Skip to content

Commit

Permalink
Remove mocks and elaborate on comments.
Browse files Browse the repository at this point in the history
  • Loading branch information
abhishekrb19 committed May 29, 2024
1 parent c6b8cc8 commit 16658e1
Show file tree
Hide file tree
Showing 5 changed files with 181 additions and 125 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -126,7 +126,7 @@ public SegmentLocalCacheManager(
);
}

if (this.config.getNumThreadsToLoadSegmentsIntoPageCacheOnDownload() > 0) {
if (config.getNumThreadsToLoadSegmentsIntoPageCacheOnDownload() > 0) {
loadOnDownloadExec = Executors.newFixedThreadPool(
config.getNumThreadsToLoadSegmentsIntoPageCacheOnDownload(),
Execs.makeThreadFactory("LoadSegmentsIntoPageCacheOnDownload-%s")
Expand All @@ -148,11 +148,11 @@ public List<DataSegment> getCachedSegments() throws IOException
"canHandleSegments() is false. getCachedSegments() must be invoked only when canHandleSegments() returns true."
);
}
final File baseDir = getInfoDir();
FileUtils.mkdirp(baseDir);
final File infoDir = getEffectiveInfoDir();
FileUtils.mkdirp(infoDir);

final List<DataSegment> cachedSegments = new ArrayList<>();
final File[] segmentsToLoad = baseDir.listFiles();
final File[] segmentsToLoad = infoDir.listFiles();

int ignored = 0;

Expand All @@ -173,7 +173,7 @@ public List<DataSegment> getCachedSegments() throws IOException
}
}
catch (Exception e) {
log.makeAlert(e, "Failed to load segment from segmentInfo file")
log.makeAlert(e, "Failed to load segment from segment cache file.")
.addData("file", file)
.emit();
}
Expand All @@ -191,8 +191,7 @@ public List<DataSegment> getCachedSegments() throws IOException
@Override
public void storeInfoFile(DataSegment segment) throws IOException
{
final File infoDir = getInfoDir();
final File segmentInfoCacheFile = new File(infoDir, segment.getId().toString());
final File segmentInfoCacheFile = new File(getEffectiveInfoDir(), segment.getId().toString());
if (!segmentInfoCacheFile.exists()) {

Check failure

Code scanning / CodeQL

Uncontrolled data used in path expression High

This path depends on a
user-provided value
.
jsonMapper.writeValue(segmentInfoCacheFile, segment);

Check failure

Code scanning / CodeQL

Uncontrolled data used in path expression High

This path depends on a
user-provided value
.
}
Expand All @@ -201,7 +200,7 @@ public void storeInfoFile(DataSegment segment) throws IOException
@Override
public void removeInfoFile(DataSegment segment)
{
final File segmentInfoCacheFile = new File(getInfoDir(), segment.getId().toString());
final File segmentInfoCacheFile = new File(getEffectiveInfoDir(), segment.getId().toString());
if (!segmentInfoCacheFile.delete()) {
log.warn("Unable to delete cache file[%s] for segment[%s].", segmentInfoCacheFile, segment.getId());
}
Expand Down Expand Up @@ -248,7 +247,19 @@ private SegmentizerFactory getSegmentFactory(final File segmentFiles) throws Seg
return factory;
}

private File getInfoDir()
/**
* Determines and returns the effective segment info directory based on the configuration settings.
* The directory is selected based on the following configurations injected into this class:
* <ul>
* <li>{@link SegmentLoaderConfig#infoDir} - If this is set, it is used as the info directory.</li>
* <li>{@link SegmentLoaderConfig#locations} - If the info directory is not set, the first location from this list is used.</li>
* <li>List of {@link StorageLocation}s injected - If both the info directory and locations list are not set, the
* first storage location is used.</li>
* </ul>
*
* @throws DruidException if none of the configurations are set, and the info directory cannot be determined.
*/
private File getEffectiveInfoDir()
{
final File infoDir;
if (config.getInfoDir() != null) {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -41,19 +41,14 @@
import org.junit.Rule;
import org.junit.Test;
import org.junit.rules.TemporaryFolder;
import org.mockito.ArgumentCaptor;
import org.mockito.Mockito;

import java.io.File;
import java.io.IOException;
import java.util.ArrayList;
import java.util.Collections;
import java.util.Comparator;
import java.util.List;
import java.util.concurrent.atomic.AtomicInteger;

import static org.mockito.ArgumentMatchers.any;

/**
* Similar to {@link SegmentLoadDropHandlerTest}. This class includes tests that cover the
* storage location layer as well.
Expand All @@ -65,7 +60,7 @@ public class SegmentLoadDropHandlerCacheTest
@Rule
public TemporaryFolder temporaryFolder = new TemporaryFolder();
private SegmentLoadDropHandler loadDropHandler;
private DataSegmentAnnouncer segmentAnnouncer;
private TestDataSegmentAnnouncer segmentAnnouncer;
private DataSegmentServerAnnouncer serverAnnouncer;
private SegmentManager segmentManager;
private SegmentLoaderConfig loaderConfig;
Expand Down Expand Up @@ -108,7 +103,7 @@ public List<StorageLocationConfig> getLocations()
objectMapper
);
segmentManager = new SegmentManager(cacheManager);
segmentAnnouncer = Mockito.mock(DataSegmentAnnouncer.class);
segmentAnnouncer = new TestDataSegmentAnnouncer();

observedAnnouncedServerCount = new AtomicInteger(0);
serverAnnouncer = new DataSegmentServerAnnouncer()
Expand Down Expand Up @@ -205,24 +200,17 @@ public void testLoadLocalCache() throws IOException, SegmentLoadingException
Assert.assertEquals(1, observedAnnouncedServerCount.get());

// Verify the expected announcements
ArgumentCaptor<Iterable<DataSegment>> argCaptor = ArgumentCaptor.forClass(Iterable.class);
Mockito.verify(segmentAnnouncer).announceSegments(argCaptor.capture());
List<DataSegment> announcedSegments = new ArrayList<>();
argCaptor.getValue().forEach(announcedSegments::add);
announcedSegments.sort(Comparator.comparing(DataSegment::getVersion));
Assert.assertEquals(expectedSegments, announcedSegments);

// make sure adding segments beyond allowed size fails
Mockito.reset(segmentAnnouncer);
Assert.assertTrue(segmentAnnouncer.getObservedSegments().containsAll(expectedSegments));

// Make sure adding segments beyond allowed size fails
DataSegment newSegment = makeSegment("test", "new-segment");
loadDropHandler.addSegment(newSegment, null);
Mockito.verify(segmentAnnouncer, Mockito.never()).announceSegment(any());
Mockito.verify(segmentAnnouncer, Mockito.never()).announceSegments(any());
Assert.assertFalse(segmentAnnouncer.getObservedSegments().contains(newSegment));

// clearing some segment should allow for new segments
// Clearing some segment should allow for new segments
loadDropHandler.removeSegment(expectedSegments.get(0), null, false);
loadDropHandler.addSegment(newSegment, null);
Mockito.verify(segmentAnnouncer).announceSegment(newSegment);
Assert.assertTrue(segmentAnnouncer.getObservedSegments().contains(newSegment));

loadDropHandler.stop();
Assert.assertEquals(0, observedAnnouncedServerCount.get());
Expand Down

0 comments on commit 16658e1

Please sign in to comment.