⚠ This page is served via a proxy. Original site: https://github.com
This service does not collect credentials or authentication data.
Skip to content

Fix MSQ compaction state and native interval locking, add test coverage#18950

Open
cecemei wants to merge 32 commits intoapache:masterfrom
cecemei:compact-test
Open

Fix MSQ compaction state and native interval locking, add test coverage#18950
cecemei wants to merge 32 commits intoapache:masterfrom
cecemei:compact-test

Conversation

@cecemei
Copy link
Contributor

@cecemei cecemei commented Jan 24, 2026

Description

This PR fixes several issues with MSQ compaction state and native compaction interval locking, and adds comprehensive test coverage for MSQ compaction tasks.

Fixed compaction state issues

  • Filter propagation: MSQ compaction runner now propagates filter to compaction state
  • Query granularity propagation: MSQ compaction runner now reads query granularity from inputs and propagates it to compaction state

Fixed native compaction interval locking

  • Native runner's index task now locks intervals from CompactionTask instead of existing segments' intervals

Added MSQCompactionTaskRunTest

  • New test suite covering MSQ compaction task execution
  • Tests various compaction scenarios including partition handling, granularity changes, and metrics aggregation

Refactored test infrastructure

  • Extracted CompactionTaskRunBase to support both native and MSQ compaction tests
  • Added NativeCompactionTaskRunTest to provide dedicated test coverage for native compaction
  • Created TestSpyTaskActionClient utility for better test observability

Added builder pattern support

  • Added toBuilder() method to CompactionState with a new Builder class
  • Added toBuilder() method to DimensionsSpec

Key changed/added classes in this PR

  • MSQCompactionTaskRunTest
  • NativeCompactionTaskRunTest
  • CompactionTaskRunBase
  • CompactionState.Builder
  • DimensionsSpec.toBuilder()
  • TestSpyTaskActionClient

This PR has:

  • been self-reviewed.
  • added documentation for new or modified features or behaviors.
  • a release note entry in the PR description.
  • added Javadocs for most classes and all non-trivial methods. Linked related entities via Javadoc links.
  • added or updated version, license, or notice information in licenses.yaml
  • added comments explaining the "why" and the intent of the code wherever would not be obvious for an unfamiliar reader.
  • added unit tests or modified existing tests to cover new code paths, ensuring the threshold for code coverage is met.
  • added integration tests.
  • been tested in a test Druid cluster.

@github-actions github-actions bot added Area - Batch Ingestion Area - Ingestion Area - MSQ For multi stage queries - https://github.com/apache/druid/issues/12262 labels Jan 24, 2026
@cecemei cecemei changed the title MSQ compaction runner run test Add MSQCompactionTaskRunTest Jan 28, 2026
Assert.assertEquals(original.isIncludeAllDimensions(), rebuilt.isIncludeAllDimensions());
Assert.assertEquals(original.useSchemaDiscovery(), rebuilt.useSchemaDiscovery());
Assert.assertEquals(original.isForceSegmentSortByTimeConfigured(), rebuilt.isForceSegmentSortByTimeConfigured());
Assert.assertEquals(original.getSpatialDimensions(), rebuilt.getSpatialDimensions());

Check notice

Code scanning / CodeQL

Deprecated method or constructor invocation Note test

Invoking
DimensionsSpec.getSpatialDimensions
should be avoided because it has been deprecated.
Assert.assertEquals(original.isIncludeAllDimensions(), rebuilt.isIncludeAllDimensions());
Assert.assertEquals(original.useSchemaDiscovery(), rebuilt.useSchemaDiscovery());
Assert.assertEquals(original.isForceSegmentSortByTimeConfigured(), rebuilt.isForceSegmentSortByTimeConfigured());
Assert.assertEquals(original.getSpatialDimensions(), rebuilt.getSpatialDimensions());

Check notice

Code scanning / CodeQL

Deprecated method or constructor invocation Note test

Invoking
DimensionsSpec.getSpatialDimensions
should be avoided because it has been deprecated.
@cecemei cecemei changed the title Add MSQCompactionTaskRunTest Fix MSQ compaction state and native interval locking, add test coverage Jan 28, 2026
@cecemei cecemei marked this pull request as ready for review January 28, 2026 05:05
Copy link
Contributor

@capistrant capistrant left a comment

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Thank you for taking the time to extend all the existing testing code to work with MSQ, this is awesome!

Left some comments, I think my main questions/concerns are around the locking change as well as MSQ and dropExisting enforcement

*/
default boolean forceDropExisting()
{
return true;
Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Was this a nasty bug that MSQ compaction didn't enforce dropExisting being true in the past? cuz like I'm sure there are people using it in production and not setting their io config explicitly.. so I guess their tasks would now fail until they set the config? I always thought dropExisting was a native only config since MSQ always creates tombstones regardless of this value. But I guess if REPLACE_LEGACY is used to make any other decisions behind the scenes it could be doing something unexpected?

Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

MSQ actually never read this config, but instead just run with dropExisting, even though it's set to false (which is also the default) in the config. I thought this would be somewhat confusing if ppl really look into what this field means.

So it's not a real change for compaction runner, the result would still be the same but instead we just want the config to be honest. I was also hesistant about this might break anything, so also considering maybe just revert this change in case it breaks something.

Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

In my test cluster, my msq compact supervisor does start failing to create tasks after deploying your feature branch. I think this would be a sad change to force people into making.

Caused by: org.apache.druid.java.util.common.IAE: Invalid config: runner[class org.apache.druid.msq.indexing.MSQCompactionRunner] must run with dropExisting

Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

yea i made the dropExisting change in another pr: https://github.com/apache/druid/pull/18968/changes#diff-280dab320bc1341105272f99d3ec8e7f8626c9d117f7affa933bc1a522bd465aL368, but the check here is probably not all that necessary. anyway i removed this check here so the task wont fail.

{
final List<DataSegment> segments = segmentProvider.findSegments(taskActionClient);
return determineLockGranularityAndTryLockWithSegments(taskActionClient, segments, segmentProvider::checkSegments);
return determineLockGranularityAndTryLock(taskActionClient, List.of(segmentProvider.interval));
Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Is this what you refer to in Native runner's index task now locks intervals from CompactionTask instead of existing segments' intervals? Isn't this code on the path for both native and MSQ compaction? that is at least how I interpret it. Not that I think that makes this wrong, but I'm trying to make sure I understand the scope of the change.

Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

an aside, I think this would make determineLockGranularityAndTryLockWithSegments unused now. do we want to leave that around for future use or remove it?

Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Yes this is on both path. This would make a different say compaction is for a day, but maybe we only have segments for 1 hour. This actually broke msq runner since the task lock is not that smart to figure out it's just the same thing, but then i felt this might be just before concurrent append & replace times that we try to lock less, we should probably move things to the concurrent append & replace world.

Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

removed determineLockGranularityAndTryLockWithSegments

Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

I am not sure if it safe to remove this. IIUC, the original logic was to fetch all the segments that overlap (even if partially) with the semgentProvider.interval, then try to lock the umbrella interval of the segments. So the final interval locked may be bigger than the original interval. The idea is that a task will be able to replace only those segments which are fully contained in the interval that is locked by that task.

@cecemei , could you share some more details on the issue that you encountered with the MSQ runner and the hour granularity? Maybe there could be an alternative solution.

Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

ah i was thinking the case when the umbrella interval is smaller than the original interval (which is the case in CompactionTaskRunBase, the segment is hourly and there's only segments in hour 0 - hour 3, so compaction task only locks for 3 hours, but the interval can be all day (see NativeCompactionTaskRunTest). MSQ runner failed when getting a task lock with the following change:

diff.patch

Cannot create a new taskLockPosse for request[TimeChunkLockRequest{lockType=REPLACE, groupId='compact_test_mgmlcmch_2026-02-03T18:37:51.661Z', dataSource='test', interval=2014-01-01T00:00:00.000Z/2014-01-02T00:00:00.000Z, preferredVersion='null', priority=25, revoked=false}] because existing locks[[TaskLockPosse{taskLock=TimeChunkLock{type=REPLACE, groupId='compact_test_mgmlcmch_2026-02-03T18:37:51.661Z', dataSource='test', interval=2014-01-01T00:00:00.000Z/2014-01-01T03:00:00.000Z, version='2026-02-03T18:37:51.673Z', priority=25, revoked=false}, taskIds=[compact_test_mgmlcmch_2026-02-03T18:37:51.661Z]}]] have same or higher priorities

Copy link
Contributor Author

@cecemei cecemei Feb 3, 2026

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

i'm not sure i quite understand the part replace only those segments which are fully contained in the locked interval? do you mean when append segments get upgrade? for compaction, i think testPartialIntervalCompactWithFinerSegmentGranularity proves that a smaller interval lock works?

cecemei and others added 5 commits February 1, 2026 22:14
…n/task/CompactionTask.java

Co-authored-by: Lucas Capistrant <capistrant@users.noreply.github.com>
…n/task/CompactionTask.java

Co-authored-by: Lucas Capistrant <capistrant@users.noreply.github.com>
Copy link
Contributor

@capistrant capistrant left a comment

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Thanks for responding to my questions @cecemei

I think this should be ready then. Could you add a release notes section to the PR description that spec's out the two bug fixes you did for the lastCompactionState in MSQ compaction. Those may be of interest to have in druid 37 release notes. I'll leave it up to you if you also think adding a dedicated note about the locking changes.

Copy link
Contributor

@kfaraz kfaraz left a comment

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Left some questions/suggestions based on first glance.

public static final Granularity FIFTEEN_MINUTE = GranularityType.FIFTEEN_MINUTE.getDefaultGranularity();
public static final Granularity THIRTY_MINUTE = GranularityType.THIRTY_MINUTE.getDefaultGranularity();
public static final Granularity HOUR = GranularityType.HOUR.getDefaultGranularity();
public static final Granularity THREE_HOUR = GranularityType.THREE_HOUR.getDefaultGranularity();
Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Why is this new granularity needed?

Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

added this for the test because compaction task automatically picks the finest gran (hourly) if it's not specified, but msq runner can only process one interval per task (there's 3 hours).

Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Let's use one of the existing standard granularities then, like SIX_HOUR or EIGHT_HOUR and update the tests accordingly. Would that work?

Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

updated to use SIX_HOUR and reverted THREE_HOUR change

{
final List<DataSegment> segments = segmentProvider.findSegments(taskActionClient);
return determineLockGranularityAndTryLockWithSegments(taskActionClient, segments, segmentProvider::checkSegments);
return determineLockGranularityAndTryLock(taskActionClient, List.of(segmentProvider.interval));
Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

I am not sure if it safe to remove this. IIUC, the original logic was to fetch all the segments that overlap (even if partially) with the semgentProvider.interval, then try to lock the umbrella interval of the segments. So the final interval locked may be bigger than the original interval. The idea is that a task will be able to replace only those segments which are fully contained in the interval that is locked by that task.

@cecemei , could you share some more details on the issue that you encountered with the MSQ runner and the hour granularity? Maybe there could be an alternative solution.

cecemei and others added 2 commits February 3, 2026 10:21
Co-authored-by: Kashif Faraz <kashif.faraz@gmail.com>
*
* @return true if aligned intervals are required by this runner, false otherwise.
*/
boolean requireAlignedInterval();
Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Nit: I wonder if we ever allow non-aligned intervals even for native compaction.
I don't see that field being set while creating auto-compaction tasks.

I feel adding this new method to the CompactionRunner interface for a field which is never used is overkill.
Let's just continue ignoring this field in the MSQ compaction runner as we do today and just log a warning message in MSQ compaction runner if it is ever set to true.

I think we will just go ahead and deprecate this field and remove it altogether in a future release.

Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Copy link
Contributor

@kfaraz kfaraz Feb 5, 2026

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

#14127

I think we have had the parameter for backwards compat for a while, we can probably deprecate it now since auto-compaction does not use it and MSQ does not use it either.

return this;
}

public TaskActionTestKit setBatchSegmentAllocation(boolean batchSegmentAllocation)
Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Just curious, have we added new tests which need this?

Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

ah i think i had some issues with batch segment allocation with segment lock, and some pending segments cache was not removed properly or something (maybe i should have written it down), and have decided to disable the batch allocation to avoid this issue

Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

yea so it's set to false in CompactionTaskRunBase

Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Oh, if that is the case, we would need to dig deeper into it since it sounds like a bug and batch segment allocation is the default. It should work for all cases.

Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

it's only for segment lock, and i wasn't sure it's due to test setup or anything.

Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Okay, in that case, let's skip the test case for segment lock and add a comment or add a test case and mark it disabled. That way, we would be aware that there is a bug and can address it later.

Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

i could not reproduce the test cases any more, maybe it disappeared after we switched to use the widen interval. anyways, so i added the segmentQueue test param.

@cecemei cecemei requested a review from kfaraz February 5, 2026 06:56
Copy link
Contributor

@kfaraz kfaraz left a comment

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Thanks for your patience, @cecemei !
I have left some final thoughts. We should be good to merge once these are addressed.

return inputSpec;
}

@Deprecated
Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Nit: Adding a short javadoc indicating why this is deprecated would be nice.

Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

added

}

@Provides
IndexerControllerContext.Builder providesContextBuilder(Injector injector)
Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

We need not use the Injector directly here.

Suggested change
IndexerControllerContext.Builder providesContextBuilder(Injector injector)
IndexerControllerContext.Builder getControllerContextBuilder(
@EscalatedGlobal ServiceClientFactory serviceClientFactory,
OverlordClient overlordClient
)

Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Also, is there any functional change here or is the code just being moved from MSQControllerTask?

Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

there's no functional change, it's just easier for testing so that we can inject the context in tests

Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

controller context need the injector... i originally added separate class as well but i think some tests breaks since the guice module asks for class instance even before it asks to provide context instance. it's not ideal but injector is in many places.

*/
public class IndexerControllerContext implements ControllerContext
{
public interface Builder
Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

It might be cleaner to move this into a separate file and call it IndexerControllerContextFactory to align with other similar factory classes in Druid.

Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

updated

this.overlordClient = overlordClient;
this.memoryIntrospector = injector.getInstance(MemoryIntrospector.class);
final StorageConnectorProvider storageConnectorProvider = injector.getInstance(Key.get(StorageConnectorProvider.class, MultiStageQuery.class));
final StorageConnectorProvider storageConnectorProvider = injector.getInstance(Key.get(
Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Suggestion: Change doesn't seem necessary. It's best to avoid formatting changes unless they really help with readability.

Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

this has been reverted

return this;
}

public TaskActionTestKit setBatchSegmentAllocation(boolean batchSegmentAllocation)
Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Okay, in that case, let's skip the test case for segment lock and add a comment or add a test case and mark it disabled. That way, we would be aware that there is a bug and can address it later.

* <p>
* Useful for verifying that tasks publish the expected segments and schemas in integration tests.
*/
public class TestSpyTaskActionClient implements TaskActionClient
Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

This action client is very specific to segment transactions and might not have a wide usage.

I would suggest the following:

  • Move this test class to CompactionTaskRunBase itself
  • Rename it to something simpler TestTaskActionClient or WrappingTaskActionClient. (The "spy" is a little misleading since it suggests some usage of Mockito spy utilities).
  • Avoid javadocs since this code is unit-test-only and seems self explanatory. Add only 1-line javadocs or regular comments where necessary.

Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

this is done

Comment on lines 133 to 138
for (LockGranularity lockGranularity : new LockGranularity[]{LockGranularity.TIME_CHUNK}) {
for (boolean useCentralizedDatasourceSchema : new boolean[]{false}) {
for (boolean useSegmentMetadataCache : new boolean[]{false, true}) {
for (boolean useConcurrentLocks : new boolean[]{false, true}) {
for (Interval inputInterval : new Interval[]{TEST_INTERVAL}) {
for (Granularity segmentGran : new Granularity[]{Granularities.SIX_HOUR}) {
Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

I think only 2 for loops (for useSegmentMetadataCache and useConcurrentLocks) should be enough here. The other parameters seem to be taking a single value only.

for (boolean useConcurrentLocks : new boolean[]{false, true}) {
for (Interval inputInterval : new Interval[]{TEST_INTERVAL}) {
for (Granularity segmentGran : new Granularity[]{Granularities.SIX_HOUR}) {
String name = StringUtils.format(
Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

This should be done in the @Paramaters annotation itself.

}

@Test
public void testWithSegmentGranularity() throws Exception
Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Is this test not needed anymore?

Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

segment granularity has been tested very throughly in this test after this change, so we dont need it any more.

import java.util.stream.Collectors;

@RunWith(Parameterized.class)
public class CompactionTaskRunTest extends IngestionTestBase
public abstract class CompactionTaskRunBase
Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

The diff for this class seems big and it is a little difficult to ensure that no assertions have actually changed.
Since we are changing some core logic in this PR, I would advise minimizing the changes to this test class so that we can be certain that the new code works correctly with all existing tests.

If any refactor is needed in this test, we can do it in a follow up PR.
For the time being, if some methods need to be reused for the MSQCompactionTaskRunTest, they may be copied over or we may make them public static where applicable.

Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

motivation to change this test is that it doesnt cover msq runner, and while re-writing it i find some tests too long to read, and some issues in msq runner have been discovered due to making it run with msq runner. the test coverage has improved a lot in this class, priori to this pr, this test only tests native compaction runner and also dont use a real task action client. i was surprised by how little test coverage we have on msq compaction runner, wanted to use this test file for any msq compaction runner related change in the future.

the changes in this pr actually was mostly not covered by this test (before this pr), only the interval locking is related. i re-ran the old test with the interval lock change, the failures are due to the interval diff in compaction state, and another test failed due to lock interval covers the input interval now. so all seems expected.

Copy link
Contributor Author

@cecemei cecemei left a comment

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Thanks for reviewing this @kfaraz ! Addressed all your comments.

return this;
}

public TaskActionTestKit setUseCentralizedDatasourceSchema(boolean useCentralizedDatasourceSchema)
Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

it's not supported by msq engine, schema is not saved


public TaskActionTestKit setBatchSegmentAllocation(boolean batchSegmentAllocation)
{
if (configFinalized.get()) {
Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

the before method was called before @test, wont that mean we can't change the config between tests, but in setup only?

}


public Builder toBuilder()
Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

ah i thought this toBuilder() is more fluent style? like:

getDefaultCompactionState().toBuilder().dimensionSpec().build()

GranularityType.HOUR,
Intervals.of("2016-06-27T01:00:00.000Z/2016-06-27T02:00:00.000Z")
Intervals.of("2016-06-27T01:00:00.000Z/2016-06-27T02:00:00.000Z"),
new CompactionTransformSpec(
Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

tests without filters are still the same (without transform spec).

} else {
// The changed granularity would result in a new virtual column that needs to be aggregated upon.
dimensionSpecs.add(new DefaultDimensionSpec(TIME_VIRTUAL_COLUMN, TIME_VIRTUAL_COLUMN, ColumnType.LONG));
if (!dataSchema.getDimensionsSpec().getDimensionNames().contains(ColumnHolder.TIME_COLUMN_NAME)) {
Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

added

import java.util.stream.Collectors;

@RunWith(Parameterized.class)
public class CompactionTaskRunTest extends IngestionTestBase
public abstract class CompactionTaskRunBase
Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

motivation to change this test is that it doesnt cover msq runner, and while re-writing it i find some tests too long to read, and some issues in msq runner have been discovered due to making it run with msq runner. the test coverage has improved a lot in this class, priori to this pr, this test only tests native compaction runner and also dont use a real task action client. i was surprised by how little test coverage we have on msq compaction runner, wanted to use this test file for any msq compaction runner related change in the future.

the changes in this pr actually was mostly not covered by this test (before this pr), only the interval locking is related. i re-ran the old test with the interval lock change, the failures are due to the interval diff in compaction state, and another test failed due to lock interval covers the input interval now. so all seems expected.

return inputSpec;
}

@Deprecated
Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

added

return this;
}

public TaskActionTestKit setBatchSegmentAllocation(boolean batchSegmentAllocation)
Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

i could not reproduce the test cases any more, maybe it disappeared after we switched to use the widen interval. anyways, so i added the segmentQueue test param.


public TaskActionTestKit setBatchSegmentAllocation(boolean batchSegmentAllocation)
{
if (configFinalized.get()) {
Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

well this is just to prevent ppl from accidently calling it inside test method

}

@Test
public void testWithSegmentGranularity() throws Exception
Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

segment granularity has been tested very throughly in this test after this change, so we dont need it any more.

@cecemei cecemei requested a review from kfaraz February 5, 2026 19:34
@capistrant capistrant self-requested a review February 5, 2026 21:12
Sign up for free to join this conversation on GitHub. Already have an account? Sign in to comment

Labels

Area - Batch Ingestion Area - Ingestion Area - MSQ For multi stage queries - https://github.com/apache/druid/issues/12262 Area - Querying Bug Release Notes

Projects

None yet

Development

Successfully merging this pull request may close these issues.

3 participants