Skip to content

Commit

Permalink
Remove config druid.coordinator.compaction.skipLockedIntervals (apa…
Browse files Browse the repository at this point in the history
…che#14807)

The value of `druid.coordinator.compaction.skipLockedIntervals` should always be `true`.
  • Loading branch information
kfaraz authored Aug 14, 2023
1 parent 0dc305f commit 786e772
Show file tree
Hide file tree
Showing 7 changed files with 32 additions and 137 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -612,7 +612,7 @@ CompactSegments initializeCompactSegmentsDuty(CompactionSegmentSearchPolicy comp
{
List<CompactSegments> compactSegmentsDutyFromCustomGroups = getCompactSegmentsDutyFromCustomGroups();
if (compactSegmentsDutyFromCustomGroups.isEmpty()) {
return new CompactSegments(config, compactionSegmentSearchPolicy, overlordClient);
return new CompactSegments(compactionSegmentSearchPolicy, overlordClient);
} else {
if (compactSegmentsDutyFromCustomGroups.size() > 1) {
log.warn(
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -131,10 +131,4 @@ public int getHttpLoadQueuePeonBatchSize()
return 1;
}

@Config("druid.coordinator.compaction.skipLockedIntervals")
public boolean getCompactionSkipLockedIntervals()
{
return true;
}

}
Original file line number Diff line number Diff line change
Expand Up @@ -48,7 +48,6 @@
import org.apache.druid.server.coordinator.CompactionStatistics;
import org.apache.druid.server.coordinator.CoordinatorCompactionConfig;
import org.apache.druid.server.coordinator.DataSourceCompactionConfig;
import org.apache.druid.server.coordinator.DruidCoordinatorConfig;
import org.apache.druid.server.coordinator.DruidCoordinatorRuntimeParams;
import org.apache.druid.server.coordinator.stats.CoordinatorRunStats;
import org.apache.druid.server.coordinator.stats.Dimension;
Expand Down Expand Up @@ -85,7 +84,6 @@ public class CompactSegments implements CoordinatorCustomDuty
status -> null != status && COMPACTION_TASK_TYPE.equals(status.getType());

private final CompactionSegmentSearchPolicy policy;
private final boolean skipLockedIntervals;
private final OverlordClient overlordClient;

// This variable is updated by the Coordinator thread executing duties and
Expand All @@ -95,23 +93,13 @@ public class CompactSegments implements CoordinatorCustomDuty
@Inject
@JsonCreator
public CompactSegments(
@JacksonInject DruidCoordinatorConfig config,
@JacksonInject CompactionSegmentSearchPolicy policy,
@JacksonInject OverlordClient overlordClient
)
{
this.policy = policy;
this.overlordClient = overlordClient;
this.skipLockedIntervals = config.getCompactionSkipLockedIntervals();
resetCompactionSnapshot();

LOG.info("Scheduling compaction with skipLockedIntervals [%s]", skipLockedIntervals);
}

@VisibleForTesting
public boolean isSkipLockedIntervals()
{
return skipLockedIntervals;
}

@VisibleForTesting
Expand Down Expand Up @@ -272,11 +260,6 @@ private Map<String, List<Interval>> getLockedIntervalsToSkip(
List<DataSourceCompactionConfig> compactionConfigs
)
{
if (!skipLockedIntervals) {
LOG.info("Not skipping any locked interval for Compaction");
return new HashMap<>();
}

final Map<String, Integer> minTaskPriority = compactionConfigs
.stream()
.collect(
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -46,7 +46,6 @@ public void testDeserialization()
Assert.assertEquals(7776000000L, config.getCoordinatorKillDurationToRetain().getMillis());
Assert.assertEquals(100, config.getCoordinatorKillMaxSegments());
Assert.assertEquals(new Duration(15 * 60 * 1000), config.getLoadTimeoutDelay());
Assert.assertTrue(config.getCompactionSkipLockedIntervals());
Assert.assertFalse(config.getCoordinatorKillIgnoreDurationToRetain());
Assert.assertEquals("http", config.getLoadQueuePeonType());

Expand All @@ -62,7 +61,6 @@ public void testDeserialization()
props.setProperty("druid.coordinator.kill.pendingSegments.on", "true");
props.setProperty("druid.coordinator.load.timeout", "PT1s");
props.setProperty("druid.coordinator.loadqueuepeon.repeatDelay", "PT0.100s");
props.setProperty("druid.coordinator.compaction.skipLockedIntervals", "false");
props.setProperty("druid.coordinator.kill.ignoreDurationToRetain", "true");

factory = Config.createFactory(props);
Expand All @@ -75,7 +73,6 @@ public void testDeserialization()
Assert.assertEquals(new Duration("PT1s"), config.getCoordinatorKillDurationToRetain());
Assert.assertEquals(10000, config.getCoordinatorKillMaxSegments());
Assert.assertEquals(new Duration("PT1s"), config.getLoadTimeoutDelay());
Assert.assertFalse(config.getCompactionSkipLockedIntervals());
Assert.assertTrue(config.getCoordinatorKillIgnoreDurationToRetain());

// Test negative druid.coordinator.kill.durationToRetain now that it is valid.
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -733,15 +733,11 @@ public void testInitializeCompactSegmentsDutyWhenCustomDutyGroupDoesNotContainsC
@Test
public void testInitializeCompactSegmentsDutyWhenCustomDutyGroupContainsCompactSegments()
{
DruidCoordinatorConfig differentConfigUsedInCustomGroup = new TestDruidCoordinatorConfig.Builder()
.withCoordinatorStartDelay(new Duration(COORDINATOR_START_DELAY))
.withCoordinatorPeriod(new Duration(COORDINATOR_PERIOD))
.withCoordinatorKillPeriod(new Duration(COORDINATOR_PERIOD))
.withCoordinatorKillMaxSegments(10)
.withCompactionSkippedLockedIntervals(false)
.withCoordinatorKillIgnoreDurationToRetain(false)
.build();
CoordinatorCustomDutyGroup compactSegmentCustomGroup = new CoordinatorCustomDutyGroup("group1", Duration.standardSeconds(1), ImmutableList.of(new CompactSegments(differentConfigUsedInCustomGroup, null, null)));
CoordinatorCustomDutyGroup compactSegmentCustomGroup = new CoordinatorCustomDutyGroup(
"group1",
Duration.standardSeconds(1),
ImmutableList.of(new CompactSegments(null, null))
);
CoordinatorCustomDutyGroups customDutyGroups = new CoordinatorCustomDutyGroups(ImmutableSet.of(compactSegmentCustomGroup));
coordinator = new DruidCoordinator(
druidCoordinatorConfig,
Expand Down Expand Up @@ -777,9 +773,6 @@ public void testInitializeCompactSegmentsDutyWhenCustomDutyGroupContainsCompactS
// CompactSegments returned by this method should be from the Custom Duty Group
CompactSegments duty = coordinator.initializeCompactSegmentsDuty(newestSegmentFirstPolicy);
Assert.assertNotNull(duty);
Assert.assertNotEquals(druidCoordinatorConfig.getCompactionSkipLockedIntervals(), duty.isSkipLockedIntervals());
// We should get the CompactSegment from the custom duty group which was created with a different config than the config in DruidCoordinator
Assert.assertEquals(differentConfigUsedInCustomGroup.getCompactionSkipLockedIntervals(), duty.isSkipLockedIntervals());
}

@Test(timeout = 3000)
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -40,7 +40,6 @@ public class TestDruidCoordinatorConfig extends DruidCoordinatorConfig
private final Duration coordinatorDatasourceKillPeriod;
private final Duration coordinatorDatasourceKillDurationToRetain;
private final int coordinatorKillMaxSegments;
private final boolean compactionSkipLockedIntervals;
private final boolean coordinatorKillIgnoreDurationToRetain;
private final String loadQueuePeonType;
private final Duration httpLoadQueuePeonRepeatDelay;
Expand All @@ -66,7 +65,6 @@ public TestDruidCoordinatorConfig(
Duration coordinatorDatasourceKillPeriod,
Duration coordinatorDatasourceKillDurationToRetain,
int coordinatorKillMaxSegments,
boolean compactionSkipLockedIntervals,
boolean coordinatorKillIgnoreDurationToRetain,
String loadQueuePeonType,
Duration httpLoadQueuePeonRepeatDelay,
Expand All @@ -92,7 +90,6 @@ public TestDruidCoordinatorConfig(
this.coordinatorDatasourceKillPeriod = coordinatorDatasourceKillPeriod;
this.coordinatorDatasourceKillDurationToRetain = coordinatorDatasourceKillDurationToRetain;
this.coordinatorKillMaxSegments = coordinatorKillMaxSegments;
this.compactionSkipLockedIntervals = compactionSkipLockedIntervals;
this.coordinatorKillIgnoreDurationToRetain = coordinatorKillIgnoreDurationToRetain;
this.loadQueuePeonType = loadQueuePeonType;
this.httpLoadQueuePeonRepeatDelay = httpLoadQueuePeonRepeatDelay;
Expand Down Expand Up @@ -203,12 +200,6 @@ public Duration getLoadTimeoutDelay()
return loadTimeoutDelay == null ? super.getLoadTimeoutDelay() : loadTimeoutDelay;
}

@Override
public boolean getCompactionSkipLockedIntervals()
{
return compactionSkipLockedIntervals;
}

@Override
public boolean getCoordinatorKillIgnoreDurationToRetain()
{
Expand Down Expand Up @@ -268,7 +259,6 @@ public static class Builder
private static final Duration DEFAULT_HTTP_LOAD_QUEUE_PEON_REPEAT_DELAY = Duration.millis(60000);
private static final Duration DEFAULT_HTTP_LOAD_QUEUE_PEON_HOST_TIMEOUT = Duration.millis(300000);
private static final int DEFAULT_HTTP_LOAD_QUEUE_PEON_BATCH_SIZE = 1;
private static final boolean DEFAULT_COMPACTION_SKIP_LOCKED_INTERVALS = true;
private static final Duration DEFAULT_COORDINATOR_AUDIT_KILL_PERIOD = new Duration("PT86400s");
private static final Duration DEFAULT_COORDINATOR_AUTIT_KILL_DURATION_TO_RETAIN = new Duration("PT7776000s");

Expand All @@ -294,7 +284,6 @@ public static class Builder
private Integer curatorLoadQueuePeonNumCallbackThreads;
private Duration httpLoadQueuePeonHostTimeout;
private Integer httpLoadQueuePeonBatchSize;
private Boolean compactionSkippedLockedIntervals;
private Duration coordinatorAuditKillPeriod;
private Duration coordinatorAuditKillDurationToRetain;

Expand Down Expand Up @@ -428,12 +417,6 @@ public Builder withHttpLoadQueuePeonBatchSize(int httpLoadQueuePeonBatchSize)
return this;
}

public Builder withCompactionSkippedLockedIntervals(boolean compactionSkippedLockedIntervals)
{
this.compactionSkippedLockedIntervals = compactionSkippedLockedIntervals;
return this;
}

public Builder withCoordianatorAuditKillPeriod(Duration coordinatorAuditKillPeriod)
{
this.coordinatorAuditKillPeriod = coordinatorAuditKillPeriod;
Expand Down Expand Up @@ -466,7 +449,6 @@ public TestDruidCoordinatorConfig build()
coordinatorDatasourceKillPeriod == null ? DEFAULT_COORDINATOR_DATASOURCE_KILL_PERIOD : coordinatorDatasourceKillPeriod,
coordinatorDatasourceKillDurationToRetain == null ? DEFAULT_COORDINATOR_DATASOURCE_KILL_DURATION_TO_RETAIN : coordinatorDatasourceKillDurationToRetain,
coordinatorKillMaxSegments == null ? DEFAULT_COORDINATOR_KILL_MAX_SEGMENTS : coordinatorKillMaxSegments,
compactionSkippedLockedIntervals == null ? DEFAULT_COMPACTION_SKIP_LOCKED_INTERVALS : compactionSkippedLockedIntervals,
coordinatorKillIgnoreDurationToRetain == null ? DEFAULT_COORDINATOR_KILL_IGNORE_DURATION_TO_RETAIN : coordinatorKillIgnoreDurationToRetain,
loadQueuePeonType == null ? DEFAULT_LOAD_QUEUE_PEON_TYPE : loadQueuePeonType,
httpLoadQueuePeonRepeatDelay == null ? DEFAULT_HTTP_LOAD_QUEUE_PEON_REPEAT_DELAY : httpLoadQueuePeonRepeatDelay,
Expand Down
Loading

0 comments on commit 786e772

Please sign in to comment.