diff --git a/.github/scripts/unit_tests_script.sh b/.github/scripts/unit_tests_script.sh index e048a19c4af3..00bcdad62622 100755 --- a/.github/scripts/unit_tests_script.sh +++ b/.github/scripts/unit_tests_script.sh @@ -21,8 +21,8 @@ unset _JAVA_OPTIONS # Set MAVEN_OPTS for Surefire launcher. MAVEN_OPTS='-Xmx2500m' ${MVN} test -pl ${MAVEN_PROJECTS} \ -${MAVEN_SKIP} -Ddruid.generic.useDefaultValueForNull=${DRUID_USE_DEFAULT_VALUE_FOR_NULL} \ --DjfrProfilerArgLine="${JFR_PROFILER_ARG_LINE}" +${MAVEN_SKIP} \ +-DjfrProfilerArgLine="${JFR_PROFILER_ARG_LINE}" -Pci sh -c "dmesg | egrep -i '(oom|out of memory|kill process|killed).*' -C 1 || exit 0" free -m ${MVN} -pl ${MAVEN_PROJECTS} jacoco:report || { echo "coverage_failure=false" >> "$GITHUB_ENV" && false; } diff --git a/.github/workflows/cron-job-its.yml b/.github/workflows/cron-job-its.yml index 31d15aaec715..d475b97bd1da 100644 --- a/.github/workflows/cron-job-its.yml +++ b/.github/workflows/cron-job-its.yml @@ -61,7 +61,7 @@ jobs: needs: build with: build_jdk: 17 - runtime_jdk: 21.0.4 + runtime_jdk: 17 testing_groups: -Dgroups=${{ matrix.testing_group }} use_indexer: middleManager group: ${{ matrix.testing_group }} @@ -75,7 +75,7 @@ jobs: needs: build with: build_jdk: 17 - runtime_jdk: 21.0.4 + runtime_jdk: 17 testing_groups: -Dgroups=${{ matrix.testing_group }} use_indexer: indexer group: ${{ matrix.testing_group }} @@ -89,7 +89,7 @@ jobs: needs: build with: build_jdk: 17 - runtime_jdk: 21.0.4 + runtime_jdk: 17 testing_groups: -Dgroups=${{ matrix.testing_group }} use_indexer: middleManager override_config_path: ./environment-configs/test-groups/prepopulated-data @@ -104,7 +104,7 @@ jobs: needs: build with: build_jdk: 17 - runtime_jdk: 21.0.4 + runtime_jdk: 17 testing_groups: -DexcludedGroups=batch-index,input-format,input-source,perfect-rollup-parallel-batch-index,kafka-index,query,query-retry,query-error,realtime-index,security,ldap-security,s3-deep-storage,gcs-deep-storage,azure-deep-storage,hdfs-deep-storage,s3-ingestion,kinesis-index,kinesis-data-format,kafka-transactional-index,kafka-index-slow,kafka-transactional-index-slow,kafka-data-format,hadoop-s3-to-s3-deep-storage,hadoop-s3-to-hdfs-deep-storage,hadoop-azure-to-azure-deep-storage,hadoop-azure-to-hdfs-deep-storage,hadoop-gcs-to-gcs-deep-storage,hadoop-gcs-to-hdfs-deep-storage,aliyun-oss-deep-storage,append-ingestion,compaction,high-availability,upgrade,shuffle-deep-store,custom-coordinator-duties use_indexer: ${{ matrix.indexer }} group: other diff --git a/.github/workflows/reusable-unit-tests.yml b/.github/workflows/reusable-unit-tests.yml index fe55e563b2dd..5f18f61eab55 100644 --- a/.github/workflows/reusable-unit-tests.yml +++ b/.github/workflows/reusable-unit-tests.yml @@ -21,11 +21,6 @@ on: required: true type: string description: 'JDK version used to test Druid' - sql_compatibility: - required: false - type: boolean - default: true - description: 'For SQL compatibility' module: required: true type: string @@ -84,13 +79,6 @@ jobs: run: | export base_ref=${{ github.base_ref }} echo "GITHUB_BASE_REF=${base_ref}" >> $GITHUB_ENV - # If sql_compatibilty is true, we want to set default_value_for_null - # which enables compatibility mode - if (${{ inputs.sql_compatibility }} == true); then - echo "DRUID_USE_DEFAULT_VALUE_FOR_NULL=false" >> $GITHUB_ENV - else - echo "DRUID_USE_DEFAULT_VALUE_FOR_NULL=true" >> $GITHUB_ENV - fi - name: test profiling run: | diff --git a/.github/workflows/revised-its.yml b/.github/workflows/revised-its.yml index 3e7394268020..bf3f45567bfb 100644 --- a/.github/workflows/revised-its.yml +++ b/.github/workflows/revised-its.yml @@ -87,7 +87,7 @@ jobs: if: ${{ needs.changes.outputs.core == 'true' || needs.changes.outputs.common-extensions == 'true' }} with: build_jdk: 17 - runtime_jdk: 21.0.4 + runtime_jdk: 17 use_indexer: middleManager script: ./it.sh github S3DeepStorage it: S3DeepStorage diff --git a/.github/workflows/static-checks.yml b/.github/workflows/static-checks.yml index 778a79db7618..6a149a46a6d3 100644 --- a/.github/workflows/static-checks.yml +++ b/.github/workflows/static-checks.yml @@ -71,7 +71,7 @@ jobs: if: ${{ matrix.java == '17' }} # errorprone requires JDK 11+ # Strict compilation requires more than 2 GB - run: ${MVN} clean -DstrictCompile compile test-compile --fail-at-end ${MAVEN_SKIP} ${MAVEN_SKIP_TESTS} + run: ${MVN} clean -DstrictCompile compile test-compile --fail-at-end ${MAVEN_SKIP} ${MAVEN_SKIP_TESTS} -T1C - name: maven install if: ${{ matrix.java == '17' }} diff --git a/.github/workflows/unit-and-integration-tests-unified.yml b/.github/workflows/unit-and-integration-tests-unified.yml index 81375f33c090..11a7d17944be 100644 --- a/.github/workflows/unit-and-integration-tests-unified.yml +++ b/.github/workflows/unit-and-integration-tests-unified.yml @@ -163,25 +163,19 @@ jobs: matrix: # Use JDK 21.0.4 to work around https://github.com/apache/druid/issues/17429 jdk: [ '11', '21.0.4' ] - name: "unit tests (jdk${{ matrix.jdk }}, sql-compat=true)" + name: "unit tests (jdk${{ matrix.jdk }})" uses: ./.github/workflows/unit-tests.yml needs: unit-tests if: ${{ always() && (needs.unit-tests.result == 'success' || needs.unit-tests.outputs.continue_tests) }} with: jdk: ${{ matrix.jdk }} - sql_compatibility: true unit-tests: - strategy: - fail-fast: false - matrix: - sql_compatibility: [ false, true ] - name: "unit tests (jdk17, sql-compat=${{ matrix.sql_compatibility }})" + name: "unit tests (jdk17)" uses: ./.github/workflows/unit-tests.yml needs: build with: jdk: 17 - sql_compatibility: ${{ matrix.sql_compatibility }} standard-its: needs: unit-tests diff --git a/.github/workflows/unit-tests.yml b/.github/workflows/unit-tests.yml index 60cf5ccbd366..42f86b48edd5 100644 --- a/.github/workflows/unit-tests.yml +++ b/.github/workflows/unit-tests.yml @@ -21,11 +21,6 @@ on: required: true type: string description: 'JDK version used to test Druid' - sql_compatibility: - required: false - type: boolean - default: true - description: 'For SQL compatibility' outputs: continue_tests: description: 'Flag to decide if next tests need to run incase coverage issue failures' @@ -72,7 +67,6 @@ jobs: uses: ./.github/workflows/reusable-unit-tests.yml with: jdk: ${{ inputs.jdk }} - sql_compatibility: ${{ inputs.sql_compatibility }} module: indexing maven_projects: 'indexing-hadoop,indexing-service,extensions-core/kafka-indexing-service,extensions-core/kinesis-indexing-service' @@ -82,7 +76,6 @@ jobs: uses: ./.github/workflows/reusable-unit-tests.yml with: jdk: ${{ inputs.jdk }} - sql_compatibility: ${{ inputs.sql_compatibility }} module: processing maven_projects: 'processing' @@ -92,7 +85,6 @@ jobs: uses: ./.github/workflows/reusable-unit-tests.yml with: jdk: ${{ inputs.jdk }} - sql_compatibility: ${{ inputs.sql_compatibility }} module: server maven_projects: 'server' @@ -100,6 +92,5 @@ jobs: uses: ./.github/workflows/reusable-unit-tests.yml with: jdk: ${{ inputs.jdk }} - sql_compatibility: ${{ inputs.sql_compatibility }} module: other maven_projects: '!processing,!indexing-hadoop,!indexing-service,!extensions-core/kafka-indexing-service,!extensions-core/kinesis-indexing-service,!server,!web-console,!integration-tests,!:druid-it-tools,!:druid-it-image,!:druid-it-cases' diff --git a/benchmarks/pom.xml b/benchmarks/pom.xml index bd8c89b4d7f6..c578cfb10bef 100644 --- a/benchmarks/pom.xml +++ b/benchmarks/pom.xml @@ -61,11 +61,6 @@ org.easymock easymock - - com.google.inject.extensions - guice-multibindings - provided - org.apache.druid druid-processing diff --git a/benchmarks/src/test/java/org/apache/druid/benchmark/FlattenJSONBenchmark.java b/benchmarks/src/test/java/org/apache/druid/benchmark/FlattenJSONBenchmark.java index 7dd0c3fc9173..ee2b39bc941a 100644 --- a/benchmarks/src/test/java/org/apache/druid/benchmark/FlattenJSONBenchmark.java +++ b/benchmarks/src/test/java/org/apache/druid/benchmark/FlattenJSONBenchmark.java @@ -68,15 +68,15 @@ public class FlattenJSONBenchmark public void prepare() throws Exception { FlattenJSONBenchmarkUtil gen = new FlattenJSONBenchmarkUtil(); - flatInputs = new ArrayList(); + flatInputs = new ArrayList<>(); for (int i = 0; i < NUM_EVENTS; i++) { flatInputs.add(gen.generateFlatEvent()); } - nestedInputs = new ArrayList(); + nestedInputs = new ArrayList<>(); for (int i = 0; i < NUM_EVENTS; i++) { nestedInputs.add(gen.generateNestedEvent()); } - jqInputs = new ArrayList(); + jqInputs = new ArrayList<>(); for (int i = 0; i < NUM_EVENTS; i++) { jqInputs.add(gen.generateNestedEvent()); // reuse the same event as "nested" } diff --git a/benchmarks/src/test/java/org/apache/druid/benchmark/GenericIndexedBenchmark.java b/benchmarks/src/test/java/org/apache/druid/benchmark/GenericIndexedBenchmark.java index 6f699efe0264..afbcb97b19f8 100644 --- a/benchmarks/src/test/java/org/apache/druid/benchmark/GenericIndexedBenchmark.java +++ b/benchmarks/src/test/java/org/apache/druid/benchmark/GenericIndexedBenchmark.java @@ -67,7 +67,7 @@ public class GenericIndexedBenchmark public static final int ITERATIONS = 10000; - static final ObjectStrategy BYTE_ARRAY_STRATEGY = new ObjectStrategy() + static final ObjectStrategy BYTE_ARRAY_STRATEGY = new ObjectStrategy<>() { @Override public Class getClazz() diff --git a/benchmarks/src/test/java/org/apache/druid/benchmark/IncrementalIndexRowTypeBenchmark.java b/benchmarks/src/test/java/org/apache/druid/benchmark/IncrementalIndexRowTypeBenchmark.java index 395f9d6d7c70..e411fc92b1ed 100644 --- a/benchmarks/src/test/java/org/apache/druid/benchmark/IncrementalIndexRowTypeBenchmark.java +++ b/benchmarks/src/test/java/org/apache/druid/benchmark/IncrementalIndexRowTypeBenchmark.java @@ -67,9 +67,9 @@ public class IncrementalIndexRowTypeBenchmark private static AggregatorFactory[] aggs; static final int DIMENSION_COUNT = 8; - private ArrayList longRows = new ArrayList(); - private ArrayList floatRows = new ArrayList(); - private ArrayList stringRows = new ArrayList(); + private ArrayList longRows = new ArrayList<>(); + private ArrayList floatRows = new ArrayList<>(); + private ArrayList stringRows = new ArrayList<>(); static { @@ -95,7 +95,7 @@ public class IncrementalIndexRowTypeBenchmark private MapBasedInputRow getLongRow(long timestamp, int dimensionCount) { Random rng = ThreadLocalRandom.current(); - List dimensionList = new ArrayList(dimensionCount); + List dimensionList = new ArrayList<>(dimensionCount); ImmutableMap.Builder builder = ImmutableMap.builder(); for (int i = 0; i < dimensionCount; i++) { String dimName = StringUtils.format("Dim_%d", i); @@ -108,7 +108,7 @@ private MapBasedInputRow getLongRow(long timestamp, int dimensionCount) private MapBasedInputRow getFloatRow(long timestamp, int dimensionCount) { Random rng = ThreadLocalRandom.current(); - List dimensionList = new ArrayList(dimensionCount); + List dimensionList = new ArrayList<>(dimensionCount); ImmutableMap.Builder builder = ImmutableMap.builder(); for (int i = 0; i < dimensionCount; i++) { String dimName = StringUtils.format("Dim_%d", i); @@ -121,7 +121,7 @@ private MapBasedInputRow getFloatRow(long timestamp, int dimensionCount) private MapBasedInputRow getStringRow(long timestamp, int dimensionCount) { Random rng = ThreadLocalRandom.current(); - List dimensionList = new ArrayList(dimensionCount); + List dimensionList = new ArrayList<>(dimensionCount); ImmutableMap.Builder builder = ImmutableMap.builder(); for (int i = 0; i < dimensionCount; i++) { String dimName = StringUtils.format("Dim_%d", i); diff --git a/benchmarks/src/test/java/org/apache/druid/benchmark/SinkQuerySegmentWalkerBenchmark.java b/benchmarks/src/test/java/org/apache/druid/benchmark/SinkQuerySegmentWalkerBenchmark.java new file mode 100644 index 000000000000..8342dd565ca1 --- /dev/null +++ b/benchmarks/src/test/java/org/apache/druid/benchmark/SinkQuerySegmentWalkerBenchmark.java @@ -0,0 +1,156 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, + * software distributed under the License is distributed on an + * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY + * KIND, either express or implied. See the License for the + * specific language governing permissions and limitations + * under the License. + */ + +package org.apache.druid.benchmark; + +import com.google.common.base.Suppliers; +import com.google.common.collect.ImmutableList; +import com.google.common.collect.ImmutableMap; +import org.apache.druid.common.config.NullHandling; +import org.apache.druid.data.input.MapBasedInputRow; +import org.apache.druid.jackson.DefaultObjectMapper; +import org.apache.druid.java.util.common.DateTimes; +import org.apache.druid.java.util.common.FileUtils; +import org.apache.druid.java.util.common.Intervals; +import org.apache.druid.java.util.common.granularity.Granularities; +import org.apache.druid.java.util.common.logger.Logger; +import org.apache.druid.java.util.emitter.core.LoggingEmitter; +import org.apache.druid.java.util.emitter.service.ServiceEmitter; +import org.apache.druid.query.Druids; +import org.apache.druid.query.QueryPlus; +import org.apache.druid.query.Result; +import org.apache.druid.query.aggregation.LongSumAggregatorFactory; +import org.apache.druid.query.context.ResponseContext; +import org.apache.druid.query.timeseries.TimeseriesQuery; +import org.apache.druid.query.timeseries.TimeseriesResultValue; +import org.apache.druid.segment.realtime.appenderator.Appenderator; +import org.apache.druid.segment.realtime.appenderator.SegmentIdWithShardSpec; +import org.apache.druid.segment.realtime.appenderator.StreamAppenderatorTester; +import org.apache.druid.segment.realtime.sink.Committers; +import org.apache.druid.timeline.partition.LinearShardSpec; +import org.openjdk.jmh.annotations.Benchmark; +import org.openjdk.jmh.annotations.BenchmarkMode; +import org.openjdk.jmh.annotations.Fork; +import org.openjdk.jmh.annotations.Level; +import org.openjdk.jmh.annotations.Measurement; +import org.openjdk.jmh.annotations.Mode; +import org.openjdk.jmh.annotations.OutputTimeUnit; +import org.openjdk.jmh.annotations.Param; +import org.openjdk.jmh.annotations.Scope; +import org.openjdk.jmh.annotations.Setup; +import org.openjdk.jmh.annotations.State; +import org.openjdk.jmh.annotations.TearDown; +import org.openjdk.jmh.annotations.Warmup; +import org.openjdk.jmh.infra.Blackhole; + +import java.io.File; +import java.util.Arrays; +import java.util.List; +import java.util.concurrent.TimeUnit; + +@State(Scope.Benchmark) +@Fork(value = 1) +@Warmup(iterations = 3) +@Measurement(iterations = 5) +@BenchmarkMode(Mode.AverageTime) +@OutputTimeUnit(TimeUnit.MILLISECONDS) +public class SinkQuerySegmentWalkerBenchmark +{ + static { + NullHandling.initializeForTests(); + } + + @Param({"10", "50", "100", "200"}) + private int numFireHydrants; + + private final LoggingEmitter loggingEmitter = new LoggingEmitter(new Logger(LoggingEmitter.class), LoggingEmitter.Level.INFO, new DefaultObjectMapper()); + private final ServiceEmitter serviceEmitter = new ServiceEmitter("test", "test", loggingEmitter); + private File cacheDir; + + private Appenderator appenderator; + + @Setup(Level.Trial) + public void setup() throws Exception + { + final String userConfiguredCacheDir = System.getProperty("druid.benchmark.cacheDir", System.getenv("DRUID_BENCHMARK_CACHE_DIR")); + cacheDir = new File(userConfiguredCacheDir); + final StreamAppenderatorTester tester = + new StreamAppenderatorTester.Builder().maxRowsInMemory(1) + .basePersistDirectory(cacheDir) + .withServiceEmitter(serviceEmitter) + .build(); + + appenderator = tester.getAppenderator(); + appenderator.startJob(); + + final SegmentIdWithShardSpec segmentIdWithShardSpec = new SegmentIdWithShardSpec( + StreamAppenderatorTester.DATASOURCE, + Intervals.of("2000/2001"), + "A", + new LinearShardSpec(0) + ); + + for (int i = 0; i < numFireHydrants; i++) { + final MapBasedInputRow inputRow = new MapBasedInputRow( + DateTimes.of("2000").getMillis(), + ImmutableList.of("dim"), + ImmutableMap.of( + "dim", + "bar_" + i, + "met", + 1 + ) + ); + appenderator.add(segmentIdWithShardSpec, inputRow, Suppliers.ofInstance(Committers.nil())); + } + } + + @TearDown(Level.Trial) + public void tearDown() throws Exception + { + appenderator.close(); + FileUtils.deleteDirectory(cacheDir); + } + + @Benchmark + @BenchmarkMode(Mode.AverageTime) + @OutputTimeUnit(TimeUnit.MILLISECONDS) + public void emitSinkMetrics(Blackhole blackhole) throws Exception + { + { + final TimeseriesQuery query1 = Druids.newTimeseriesQueryBuilder() + .dataSource(StreamAppenderatorTester.DATASOURCE) + .intervals(ImmutableList.of(Intervals.of("2000/2001"))) + .aggregators( + Arrays.asList( + new LongSumAggregatorFactory("count", "count"), + new LongSumAggregatorFactory("met", "met") + ) + ) + .granularity(Granularities.DAY) + .build(); + + final List> results = + QueryPlus.wrap(query1).run(appenderator, ResponseContext.createEmpty()).toList(); + blackhole.consume(results); + + serviceEmitter.flush(); + } + } +} diff --git a/benchmarks/src/test/java/org/apache/druid/benchmark/StupidPoolConcurrencyBenchmark.java b/benchmarks/src/test/java/org/apache/druid/benchmark/StupidPoolConcurrencyBenchmark.java index b7ed8eade071..8462a3cca58a 100644 --- a/benchmarks/src/test/java/org/apache/druid/benchmark/StupidPoolConcurrencyBenchmark.java +++ b/benchmarks/src/test/java/org/apache/druid/benchmark/StupidPoolConcurrencyBenchmark.java @@ -44,7 +44,7 @@ public static class BenchmarkPool private final AtomicLong numPools = new AtomicLong(0L); private final NonBlockingPool pool = new StupidPool<>( "simpleObject pool", - new Supplier() + new Supplier<>() { @Override public Object get() diff --git a/benchmarks/src/test/java/org/apache/druid/benchmark/indexing/OnheapIncrementalIndexBenchmark.java b/benchmarks/src/test/java/org/apache/druid/benchmark/indexing/OnheapIncrementalIndexBenchmark.java index 7c67e7895ab5..8f662f273a34 100644 --- a/benchmarks/src/test/java/org/apache/druid/benchmark/indexing/OnheapIncrementalIndexBenchmark.java +++ b/benchmarks/src/test/java/org/apache/druid/benchmark/indexing/OnheapIncrementalIndexBenchmark.java @@ -116,7 +116,7 @@ public class OnheapIncrementalIndexBenchmark private static MapBasedInputRow getLongRow(long timestamp, int rowID, int dimensionCount) { - List dimensionList = new ArrayList(dimensionCount); + List dimensionList = new ArrayList<>(dimensionCount); ImmutableMap.Builder builder = ImmutableMap.builder(); for (int i = 0; i < dimensionCount; i++) { String dimName = StringUtils.format("Dim_%d", i); diff --git a/benchmarks/src/test/java/org/apache/druid/benchmark/sequences/MergeSequenceBenchmark.java b/benchmarks/src/test/java/org/apache/druid/benchmark/sequences/MergeSequenceBenchmark.java index 995163077e50..61bee5d59332 100644 --- a/benchmarks/src/test/java/org/apache/druid/benchmark/sequences/MergeSequenceBenchmark.java +++ b/benchmarks/src/test/java/org/apache/druid/benchmark/sequences/MergeSequenceBenchmark.java @@ -80,14 +80,14 @@ public void setup() public void mergeHierarchical(Blackhole blackhole) { Iterator> iterator = sequences.iterator(); - List> partialMerged = new ArrayList>(); - List> toMerge = new ArrayList>(); + List> partialMerged = new ArrayList<>(); + List> toMerge = new ArrayList<>(); while (iterator.hasNext()) { toMerge.add(iterator.next()); if (toMerge.size() == mergeAtOnce) { - partialMerged.add(new MergeSequence(Ordering.natural(), Sequences.simple(toMerge))); - toMerge = new ArrayList>(); + partialMerged.add(new MergeSequence<>(Ordering.natural(), Sequences.simple(toMerge))); + toMerge = new ArrayList<>(); } } diff --git a/codestyle/spotbugs-exclude.xml b/codestyle/spotbugs-exclude.xml index 9a2ba6ecf1c9..9776e554b8e7 100644 --- a/codestyle/spotbugs-exclude.xml +++ b/codestyle/spotbugs-exclude.xml @@ -137,15 +137,9 @@ - - - - - diff --git a/distribution/docker/Dockerfile b/distribution/docker/Dockerfile index 0ac517a29ba1..6b16ec2b94f3 100644 --- a/distribution/docker/Dockerfile +++ b/distribution/docker/Dockerfile @@ -23,7 +23,7 @@ ARG JDK_VERSION=17 # This is because it's not able to build the distribution on arm64 due to dependency problem of web-console. See: https://github.com/apache/druid/issues/13012 # Since only java jars are shipped in the final image, it's OK to build the distribution on x64. # Once the web-console dependency problem is resolved, we can remove the --platform directive. -FROM --platform=linux/amd64 maven:3.8.4-openjdk-17-slim as builder +FROM --platform=linux/amd64 maven:3.9 as builder # Rebuild from source in this stage # This can be unset if the tarball was already built outside of Docker diff --git a/docs/configuration/index.md b/docs/configuration/index.md index df14614d95e2..26601c6db40f 100644 --- a/docs/configuration/index.md +++ b/docs/configuration/index.md @@ -890,13 +890,13 @@ These Coordinator static configurations can be defined in the `coordinator/runti |`druid.coordinator.startDelay`|The operation of the Coordinator works on the assumption that it has an up-to-date view of the state of the world when it runs, the current ZooKeeper interaction code, however, is written in a way that doesn’t allow the Coordinator to know for a fact that it’s done loading the current state of the world. This delay is a hack to give it enough time to believe that it has all the data.|`PT300S`| |`druid.coordinator.load.timeout`|The timeout duration for when the Coordinator assigns a segment to a Historical service.|`PT15M`| |`druid.coordinator.kill.pendingSegments.on`|Boolean flag for whether or not the Coordinator clean up old entries in the `pendingSegments` table of metadata store. If set to true, Coordinator will check the created time of most recently complete task. If it doesn't exist, it finds the created time of the earliest running/pending/waiting tasks. Once the created time is found, then for all datasources not in the `killPendingSegmentsSkipList` (see [Dynamic configuration](#dynamic-configuration)), Coordinator will ask the Overlord to clean up the entries 1 day or more older than the found created time in the `pendingSegments` table. This will be done periodically based on `druid.coordinator.period.indexingPeriod` specified.|true| -|`druid.coordinator.kill.on`|Boolean flag for whether or not the Coordinator should submit kill task for unused segments, that is, permanently delete them from metadata store and deep storage. If set to true, then for all whitelisted datasources (or optionally all), Coordinator will submit tasks periodically based on `period` specified. A whitelist can be set via dynamic configuration `killDataSourceWhitelist` described later.

When `druid.coordinator.kill.on` is true, segments are eligible for permanent deletion once their data intervals are older than `druid.coordinator.kill.durationToRetain` relative to the current time. If a segment's data interval is older than this threshold at the time it is marked unused, it is eligible for permanent deletion immediately after being marked unused.|false| +|`druid.coordinator.kill.on`|Boolean flag to enable the Coordinator to submit a kill task for unused segments and delete them permanently from the metadata store and deep storage.|false| |`druid.coordinator.kill.period`| The frequency of sending kill tasks to the indexing service. The value must be greater than or equal to `druid.coordinator.period.indexingPeriod`. Only applies if kill is turned on.|Same as `druid.coordinator.period.indexingPeriod`| -|`druid.coordinator.kill.durationToRetain`|Only applies if you set `druid.coordinator.kill.on` to `true`. This value is ignored if `druid.coordinator.kill.ignoreDurationToRetain` is `true`. Valid configurations must be a ISO8601 period. Druid will not kill unused segments whose interval end date is beyond `now - durationToRetain`. `durationToRetain` can be a negative ISO8601 period, which would result in `now - durationToRetain` to be in the future.

Note that the `durationToRetain` parameter applies to the segment interval, not the time that the segment was last marked unused. For example, if `durationToRetain` is set to `P90D`, then a segment for a time chunk 90 days in the past is eligible for permanent deletion immediately after being marked unused.|`P90D`| +|`druid.coordinator.kill.durationToRetain`|Duration, in ISO 8601 format, relative to the current time that identifies the data interval of segments to retain. When `druid.coordinator.kill.on` is true, any segment with a data interval ending before `now - durationToRetain` is eligible for permanent deletion. For example, if `durationToRetain` is set to `P90D`, unused segments with time intervals ending 90 days in the past are eligible for deletion. If `durationToRetain` is set to a negative ISO 8601 period, segments with future intervals ending before `now - durationToRetain` are also eligible for deletion.|`P90D`| |`druid.coordinator.kill.ignoreDurationToRetain`|A way to override `druid.coordinator.kill.durationToRetain` and tell the coordinator that you do not care about the end date of unused segment intervals when it comes to killing them. If true, the coordinator considers all unused segments as eligible to be killed.|false| |`druid.coordinator.kill.bufferPeriod`|The amount of time that a segment must be unused before it is able to be permanently removed from metadata and deep storage. This can serve as a buffer period to prevent data loss if data ends up being needed after being marked unused.|`P30D`| |`druid.coordinator.kill.maxSegments`|The number of unused segments to kill per kill task. This number must be greater than 0. This only applies when `druid.coordinator.kill.on=true`.|100| -|`druid.coordinator.balancer.strategy`|Specify the type of balancing strategy for the Coordinator to use to distribute segments among the Historical services. `diskNormalized` weights the costs according to the servers' disk usage ratios - there are known issues with this strategy distributing segments unevenly across the cluster. `random` distributes segments among services randomly.|`cost`| +|`druid.coordinator.balancer.strategy`|The [balancing strategy](../design/coordinator.md#balancing-segments-in-a-tier) used by the Coordinator to distribute segments among the Historical servers in a tier. The `cost` strategy distributes segments by minimizing a cost function, `diskNormalized` weights these costs with the disk usage ratios of the servers and `random` distributes segments randomly.|`cost`| |`druid.coordinator.loadqueuepeon.http.repeatDelay`|The start and repeat delay (in milliseconds) for the load queue peon, which manages the load/drop queue of segments for any server.|1 minute| |`druid.coordinator.loadqueuepeon.http.batchSize`|Number of segment load/drop requests to batch in one HTTP request. Note that it must be smaller than `druid.segmentCache.numLoadingThreads` config on Historical service.|1| |`druid.coordinator.asOverlord.enabled`|Boolean value for whether this Coordinator service should act like an Overlord as well. This configuration allows users to simplify a Druid cluster by not having to deploy any standalone Overlord services. If set to true, then Overlord console is available at `http://coordinator-host:port/console.html` and be sure to set `druid.coordinator.asOverlord.overlordService` also.|false| @@ -957,7 +957,7 @@ The following table shows the dynamic configuration properties for the Coordinat |`replicantLifetime`|The maximum number of Coordinator runs for which a segment can wait in the load queue of a Historical before Druid raises an alert.|15| |`replicationThrottleLimit`|The maximum number of segment replicas that can be assigned to a historical tier in a single Coordinator run. This property prevents Historical services from becoming overwhelmed when loading extra replicas of segments that are already available in the cluster.|500| |`balancerComputeThreads`|Thread pool size for computing moving cost of segments during segment balancing. Consider increasing this if you have a lot of segments and moving segments begins to stall.|`num_cores` / 2| -|`killDataSourceWhitelist`|List of specific data sources for which kill tasks are sent if property `druid.coordinator.kill.on` is true. This can be a list of comma-separated data source names or a JSON array.|none| +|`killDataSourceWhitelist`|List of specific data sources for which kill tasks can be issued if `druid.coordinator.kill.on` is true. It can be a comma-separated list of data source names or a JSON array. If `killDataSourceWhitelist` is empty, the Coordinator issues kill tasks for all data sources.|none| |`killTaskSlotRatio`|Ratio of total available task slots, including autoscaling if applicable that will be allowed for kill tasks. This value must be between 0 and 1. Only applicable for kill tasks that are spawned automatically by the coordinator's auto kill duty, which is enabled when `druid.coordinator.kill.on` is true.|0.1| |`maxKillTaskSlots`|Maximum number of tasks that will be allowed for kill tasks. This limit only applies for kill tasks that are spawned automatically by the coordinator's auto kill duty, which is enabled when `druid.coordinator.kill.on` is true.|`Integer.MAX_VALUE` - no limit| |`killPendingSegmentsSkipList`|List of data sources for which pendingSegments are _NOT_ cleaned up if property `druid.coordinator.kill.pendingSegments.on` is true. This can be a list of comma-separated data sources or a JSON array.|none| diff --git a/docs/data-management/compaction.md b/docs/data-management/compaction.md index 91ebf0d521ef..51bf7ee86427 100644 --- a/docs/data-management/compaction.md +++ b/docs/data-management/compaction.md @@ -78,7 +78,7 @@ Unless you modify the segment granularity in [`granularitySpec`](manual-compacti If segments have different segment granularities before compaction but there is some overlap in interval, Druid attempts find start and end of the overlapping interval and uses the closest segment granularity level for the compacted segment. -For example consider two overlapping segments: segment "A" for the interval 01/01/2021-01/02/2021 with day granularity and segment "B" for the interval 01/01/2021-02/01/2021. Druid attempts to combine and compact the overlapped segments. In this example, the earliest start time for the two segments is 01/01/2020 and the latest end time of the two segments is 02/01/2020. Druid compacts the segments together even though they have different segment granularity. Druid uses month segment granularity for the newly compacted segment even though segment A's original segment granularity was DAY. +For example consider two overlapping segments: segment "A" for the interval 01/01/2020-01/02/2020 with day granularity and segment "B" for the interval 01/01/2020-02/01/2020. Druid attempts to combine and compact the overlapped segments. In this example, the earliest start time for the two segments is 01/01/2020 and the latest end time of the two segments is 02/01/2020. Druid compacts the segments together even though they have different segment granularity. Druid uses month segment granularity for the newly compacted segment even though segment A's original segment granularity was day granularity. ### Query granularity handling diff --git a/docs/design/coordinator.md b/docs/design/coordinator.md index 9cb2279d4976..bc4c5ebc1cba 100644 --- a/docs/design/coordinator.md +++ b/docs/design/coordinator.md @@ -79,11 +79,19 @@ On each run, the Coordinator determines and cleans up unneeded eternity tombston ## Segment availability -If a Historical service restarts or becomes unavailable for any reason, the Coordinator will notice a service has gone missing and treat all segments served by that service as being dropped. Given a sufficient period of time, the segments may be reassigned to other Historical services in the cluster. However, each segment that is dropped is not immediately forgotten. Instead, there is a transitional data structure that stores all dropped segments with an associated lifetime. The lifetime represents a period of time in which the Coordinator will not reassign a dropped segment. Hence, if a Historical service becomes unavailable and available again within a short period of time, the Historical service will start up and serve segments from its cache without any those segments being reassigned across the cluster. +If a Historical service restarts or becomes unavailable for any reason, the Coordinator notices that a service has gone missing and treats all segments served by that service as being dropped. The segments are then reassigned to other Historical services in the cluster. However, each segment that is dropped is not immediately forgotten. Instead, there is a transitional data structure that stores all dropped segments with an associated lifetime. The lifetime represents a period of time in which the Coordinator will not reassign a dropped segment. Hence, if a Historical service becomes unavailable and available again within a short period of time, the Historical service will start up and serve segments from its cache without any of those segments being reassigned across the cluster. -## Balancing segment load +## Balancing segments in a tier -To ensure an even distribution of segments across Historical services in the cluster, the Coordinator service will find the total size of all segments being served by every Historical service each time the Coordinator runs. For every Historical service tier in the cluster, the Coordinator service will determine the Historical service with the highest utilization and the Historical service with the lowest utilization. The percent difference in utilization between the two services is computed, and if the result exceeds a certain threshold, a number of segments will be moved from the highest utilized service to the lowest utilized service. There is a configurable limit on the number of segments that can be moved from one service to another each time the Coordinator runs. Segments to be moved are selected at random and only moved if the resulting utilization calculation indicates the percentage difference between the highest and lowest servers has decreased. +Druid queries perform optimally when segments are distributed evenly across Historical services. An ideal distribution would ensure that all Historicals participate equally in the query load thus avoiding hot-spots in the system. To some extent, this can be achieved by keeping multiple replicas of a segment in a cluster. +But in a tier with several Historicals (or a low replication factor), segment replication is not sufficient to attain balance. +Thus, the Coordinator constantly monitors the set of segments present on each Historical in a tier and employs one of the following strategies to identify segments that may be moved from one Historical to another to retain balance. + +- `cost` (default): For a given segment in a tier, this strategy picks the server with the minimum "cost" of placing that segment. The cost is a function of the data interval of the segment and the data intervals of all the segments already present on the candidate server. In essence, this strategy tries to avoid placing segments with adjacent or overlapping data intervals on the same server. This is based on the premise that adjacent-interval segments are more likely to be used together in a query and placing them on the same server may lead to skewed CPU usages of Historicals. +- `diskNormalized`: A derivative of the `cost` strategy that weights the cost of placing a segment on a server with the disk usage ratio of the server. There are known issues with this strategy and is not recommended for a production cluster. +- `random`: Distributes segments randomly across servers. This is an experimental strategy and is not recommended for a production cluster. + +All of the above strategies prioritize moving segments from the Historical with the least available disk space. ## Automatic compaction diff --git a/docs/development/extensions-core/bloom-filter.md b/docs/development/extensions-core/bloom-filter.md index 30cebeef6c8c..c0167e446da4 100644 --- a/docs/development/extensions-core/bloom-filter.md +++ b/docs/development/extensions-core/bloom-filter.md @@ -23,28 +23,25 @@ title: "Bloom Filter" --> -To use this Apache Druid extension, [include](../../configuration/extensions.md#loading-extensions) `druid-bloom-filter` in the extensions load list. +To use the Apache Druid® Bloom filter extension, include `druid-bloom-filter` in the extensions load list. See [Loading extensions](../../configuration/extensions.md#loading-extensions) for more information. -This extension adds the ability to both construct bloom filters from query results, and filter query results by testing -against a bloom filter. A Bloom filter is a probabilistic data structure for performing a set membership check. A bloom -filter is a good candidate to use with Druid for cases where an explicit filter is impossible, e.g. filtering a query +This extension adds the abilities to construct Bloom filters from query results and to filter query results by testing +against a Bloom filter. A Bloom filter is a probabilistic data structure to check for set membership. A Bloom +filter is a good candidate to use when an explicit filter is impossible, such as filtering a query against a set of millions of values. Following are some characteristics of Bloom filters: -- Bloom filters are highly space efficient when compared to using a HashSet. -- Because of the probabilistic nature of bloom filters, false positive results are possible (element was not actually -inserted into a bloom filter during construction, but `test()` says true) -- False negatives are not possible (if element is present then `test()` will never say false). -- The false positive probability of this implementation is currently fixed at 5%, but increasing the number of entries -that the filter can hold can decrease this false positive rate in exchange for overall size. -- Bloom filters are sensitive to number of elements that will be inserted in the bloom filter. During the creation of bloom filter expected number of entries must be specified. If the number of insertions exceed - the specified initial number of entries then false positive probability will increase accordingly. +- Bloom filters are significantly more space efficient than HashSets. +- Because they are probabilistic, false positive results are possible with Bloom filters. For example, the `test()` function might return `true` for an element that is not within the filter. +- False negatives are not possible. If an element is present, `test()` always returns `true`. +- The false positive probability of this implementation is fixed at 5%. Increasing the number of entries that the filter can hold can decrease this false positive rate in exchange for overall size. +- Bloom filters are sensitive to the number of inserted elements. You must specify the expected number of entries at creation time. If the number of insertions exceeds the specified number of entries, the false positive probability increases accordingly. -This extension is currently based on `org.apache.hive.common.util.BloomKFilter` from `hive-storage-api`. Internally, +This extension is based on `org.apache.hive.common.util.BloomKFilter` from `hive-storage-api`. Internally, this implementation uses Murmur3 as the hash algorithm. -To construct a BloomKFilter externally with Java to use as a filter in a Druid query: +The following Java example shows how to construct a BloomKFilter externally: ```java BloomKFilter bloomFilter = new BloomKFilter(1500); @@ -56,11 +53,12 @@ BloomKFilter.serialize(byteArrayOutputStream, bloomFilter); String base64Serialized = Base64.encodeBase64String(byteArrayOutputStream.toByteArray()); ``` -This string can then be used in the native or SQL Druid query. +You can then use the Base64 encoded string in JSON-based or SQL-based queries in Druid. -## Filtering queries with a Bloom Filter +## Filter queries with a Bloom filter + +### JSON specification -### JSON Specification of Bloom Filter ```json { "type" : "bloom", @@ -70,50 +68,46 @@ This string can then be used in the native or SQL Druid query. } ``` -|Property |Description |required? | -|-------------------------|------------------------------|----------------------------------| -|`type` |Filter Type. Should always be `bloom`|yes| -|`dimension` |The dimension to filter over. | yes | -|`bloomKFilter` |Base64 encoded Binary representation of `org.apache.hive.common.util.BloomKFilter`| yes | -|`extractionFn`|[Extraction function](../../querying/dimensionspecs.md#extraction-functions) to apply to the dimension values |no| - +|Property|Description|Required| +|--------|-----------|--------| +|`type`|Filter type. Set to `bloom`.|Yes| +|`dimension`|Dimension to filter over.|Yes| +|`bloomKFilter`|Base64 encoded binary representation of `org.apache.hive.common.util.BloomKFilter`.|Yes| +|`extractionFn`|[Extraction function](../../querying/dimensionspecs.md#extraction-functions) to apply to the dimension values.|No| -### Serialized Format for BloomKFilter +### Serialized format for BloomKFilter - Serialized BloomKFilter format: +Serialized BloomKFilter format: - - 1 byte for the number of hash functions. - - 1 big endian int(That is how OutputStream works) for the number of longs in the bitset - - big endian longs in the BloomKFilter bitset +- 1 byte for the number of hash functions. +- 1 big-endian integer for the number of longs in the bitset. +- Big-endian longs in the BloomKFilter bitset. -Note: `org.apache.hive.common.util.BloomKFilter` provides a serialize method which can be used to serialize bloom filters to outputStream. +`org.apache.hive.common.util.BloomKFilter` provides a method to serialize Bloom filters to `outputStream`. -### Filtering SQL Queries +### Filter SQL queries -Bloom filters can be used in SQL `WHERE` clauses via the `bloom_filter_test` operator: +You can use Bloom filters in SQL `WHERE` clauses with the `bloom_filter_test` operator: ```sql SELECT COUNT(*) FROM druid.foo WHERE bloom_filter_test(, '') ``` -### Expression and Virtual Column Support +### Expression and virtual column support -The bloom filter extension also adds a bloom filter [Druid expression](../../querying/math-expr.md) which shares syntax +The Bloom filter extension also adds a Bloom filter [Druid expression](../../querying/math-expr.md) which shares syntax with the SQL operator. ```sql bloom_filter_test(, '') ``` -## Bloom Filter Query Aggregator +## Bloom filter query aggregator -Input for a `bloomKFilter` can also be created from a druid query with the `bloom` aggregator. Note that it is very -important to set a reasonable value for the `maxNumEntries` parameter, which is the maximum number of distinct entries -that the bloom filter can represent without increasing the false positive rate. It may be worth performing a query using -one of the unique count sketches to calculate the value for this parameter in order to build a bloom filter appropriate -for the query. +You can create an input for a `BloomKFilter` from a Druid query with the `bloom` aggregator. Make sure to set a reasonable value for the `maxNumEntries` parameter to specify the maximum number of distinct entries that the Bloom filter can represent without increasing the false positive rate. Try performing a query using +one of the unique count sketches to calculate the value for this parameter to build a Bloom filter appropriate for the query. -### JSON Specification of Bloom Filter Aggregator +### JSON specification ```json { @@ -124,15 +118,17 @@ for the query. } ``` -|Property |Description |required? | -|-------------------------|------------------------------|----------------------------------| -|`type` |Aggregator Type. Should always be `bloom`|yes| -|`name` |Output field name |yes| -|`field` |[DimensionSpec](../../querying/dimensionspecs.md) to add to `org.apache.hive.common.util.BloomKFilter` | yes | -|`maxNumEntries` |Maximum number of distinct values supported by `org.apache.hive.common.util.BloomKFilter`, default `1500`| no | +|Property|Description|Required| +|--------|-----------|--------| +|`type`|Aggregator type. Set to `bloom`.|Yes| +|`name`|Output field name.|Yes| +|`field`|[DimensionSpec](../../querying/dimensionspecs.md) to add to `org.apache.hive.common.util.BloomKFilter`.|Yes| +|`maxNumEntries`|Maximum number of distinct values supported by `org.apache.hive.common.util.BloomKFilter`. Defaults to `1500`.|No| ### Example +The following example shows a timeseries query object with a `bloom` aggregator: + ```json { "queryType": "timeseries", @@ -154,25 +150,26 @@ for the query. } ``` -response +Example response: ```json -[{"timestamp":"2015-09-12T00:00:00.000Z","result":{"userBloom":"BAAAJhAAAA..."}}] +[ + { + "timestamp":"2015-09-12T00:00:00.000Z", + "result":{"userBloom":"BAAAJhAAAA..."} + } +] ``` -These values can then be set in the filter specification described above. - -Ordering results by a bloom filter aggregator, for example in a TopN query, will perform a comparatively expensive -linear scan _of the filter itself_ to count the number of set bits as a means of approximating how many items have been -added to the set. As such, ordering by an alternate aggregation is recommended if possible. +We recommend ordering by an alternative aggregation method instead of ordering results by a Bloom filter aggregator. +Ordering results by a Bloom filter aggregator can be resource-intensive because Druid performs an expensive linear scan of the filter to approximate the count of items added to the set by counting the number of set bits. +### SQL Bloom filter aggregator -### SQL Bloom Filter Aggregator -Bloom filters can be computed in SQL expressions with the `bloom_filter` aggregator: +You can compute Bloom filters in SQL expressions with the BLOOM_FILTER aggregator. For example: ```sql SELECT BLOOM_FILTER(, ) FROM druid.foo WHERE dim2 = 'abc' ``` -but requires the setting `druid.sql.planner.serializeComplexValues` to be set to `true`. Bloom filter results in a SQL - response are serialized into a base64 string, which can then be used in subsequent queries as a filter. +Druid serializes Bloom filter results in a SQL response into a Base64 string. You can use the resulting string in subsequent queries as a filter. diff --git a/docs/development/extensions-core/druid-lookups.md b/docs/development/extensions-core/druid-lookups.md index 06283ec4d722..e3514a0d0c2d 100644 --- a/docs/development/extensions-core/druid-lookups.md +++ b/docs/development/extensions-core/druid-lookups.md @@ -81,10 +81,22 @@ This example demonstrates a polling cache that will update its on-heap cache eve ```json { - "type":"pollingLookup", - "pollPeriod":"PT10M", - "dataFetcher":{ "type":"jdbcDataFetcher", "connectorConfig":"jdbc://mysql://localhost:3306/my_data_base", "table":"lookup_table_name", "keyColumn":"key_column_name", "valueColumn": "value_column_name"}, - "cacheFactory":{"type":"onHeapPolling"} + "type": "pollingLookup", + "pollPeriod": "PT10M", + "dataFetcher": { + "type": "jdbcDataFetcher", + "connectorConfig": { + "connectURI": "jdbc://mysql://localhost:3306/my_data_base", + "user": "druid", + "password": "druid" + }, + "table": "lookup_table_name", + "keyColumn": "key_column_name", + "valueColumn": "value_column_name" + }, + "cacheFactory": { + "type": "onHeapPolling" + } } ``` @@ -94,9 +106,21 @@ This example demonstrates an off-heap lookup that will be cached once and never ```json { - "type":"pollingLookup", - "dataFetcher":{ "type":"jdbcDataFetcher", "connectorConfig":"jdbc://mysql://localhost:3306/my_data_base", "table":"lookup_table_name", "keyColumn":"key_column_name", "valueColumn": "value_column_name"}, - "cacheFactory":{"type":"offHeapPolling"} + "type": "pollingLookup", + "dataFetcher": { + "type": "jdbcDataFetcher", + "connectorConfig": { + "connectURI": "jdbc://mysql://localhost:3306/my_data_base", + "user": "druid", + "password": "druid" + }, + "table": "lookup_table_name", + "keyColumn": "key_column_name", + "valueColumn": "value_column_name" + }, + "cacheFactory": { + "type": "offHeapPolling" + } } ``` @@ -125,10 +149,27 @@ Guava cache configuration spec. ```json { - "type":"loadingLookup", - "dataFetcher":{ "type":"jdbcDataFetcher", "connectorConfig":"jdbc://mysql://localhost:3306/my_data_base", "table":"lookup_table_name", "keyColumn":"key_column_name", "valueColumn": "value_column_name"}, - "loadingCacheSpec":{"type":"guava"}, - "reverseLoadingCacheSpec":{"type":"guava", "maximumSize":500000, "expireAfterAccess":100000, "expireAfterWrite":10000} + "type": "loadingLookup", + "dataFetcher": { + "type": "jdbcDataFetcher", + "connectorConfig": { + "connectURI": "jdbc://mysql://localhost:3306/my_data_base", + "user": "druid", + "password": "druid" + }, + "table": "lookup_table_name", + "keyColumn": "key_column_name", + "valueColumn": "value_column_name" + }, + "loadingCacheSpec": { + "type": "guava" + }, + "reverseLoadingCacheSpec": { + "type": "guava", + "maximumSize": 500000, + "expireAfterAccess": 100000, + "expireAfterWrite": 10000 + } } ``` @@ -146,10 +187,28 @@ Off heap cache is backed by [MapDB](http://www.mapdb.org/) implementation. MapDB ```json { - "type":"loadingLookup", - "dataFetcher":{ "type":"jdbcDataFetcher", "connectorConfig":"jdbc://mysql://localhost:3306/my_data_base", "table":"lookup_table_name", "keyColumn":"key_column_name", "valueColumn": "value_column_name"}, - "loadingCacheSpec":{"type":"mapDb", "maxEntriesSize":100000}, - "reverseLoadingCacheSpec":{"type":"mapDb", "maxStoreSize":5, "expireAfterAccess":100000, "expireAfterWrite":10000} + "type": "loadingLookup", + "dataFetcher": { + "type": "jdbcDataFetcher", + "connectorConfig": { + "connectURI": "jdbc://mysql://localhost:3306/my_data_base", + "user": "druid", + "password": "druid" + }, + "table": "lookup_table_name", + "keyColumn": "key_column_name", + "valueColumn": "value_column_name" + }, + "loadingCacheSpec": { + "type": "mapDb", + "maxEntriesSize": 100000 + }, + "reverseLoadingCacheSpec": { + "type": "mapDb", + "maxStoreSize": 5, + "expireAfterAccess": 100000, + "expireAfterWrite": 10000 + } } ``` diff --git a/docs/ingestion/partitioning.md b/docs/ingestion/partitioning.md index d1501f5fe0ff..22fb253314c3 100644 --- a/docs/ingestion/partitioning.md +++ b/docs/ingestion/partitioning.md @@ -81,8 +81,8 @@ The following table describes how to configure sorting. |Method|Configuration| |------|------------| |[SQL](../multi-stage-query/index.md)|Uses order of fields in [`CLUSTERED BY`](../multi-stage-query/concepts.md#clustering) or [`segmentSortOrder`](../multi-stage-query/reference.md#context) in the query context| -|[Kafka](../ingestion/kafka-ingestion.md) or [Kinesis](../ingestion/kinesis-ingestion.md)|Uses order of fields in [`dimensionsSpec`](ingestion-spec.md#granularityspec)| -|[Native batch](native-batch.md) or [Hadoop](hadoop.md)|Uses order of fields in [`dimensionsSpec`](ingestion-spec.md#granularityspec)| +|[Kafka](../ingestion/kafka-ingestion.md) or [Kinesis](../ingestion/kinesis-ingestion.md)|Uses order of fields in [`dimensionsSpec`](ingestion-spec.md#dimensionsspec)| +|[Native batch](native-batch.md) or [Hadoop](hadoop.md)|Uses order of fields in [`dimensionsSpec`](ingestion-spec.md#dimensionsspec)| :::info Druid implicitly sorts rows within a segment by `__time` first before any `dimensions` or `CLUSTERED BY` fields, unless diff --git a/docs/ingestion/tasks.md b/docs/ingestion/tasks.md index 92f21b924874..9396378bcdb5 100644 --- a/docs/ingestion/tasks.md +++ b/docs/ingestion/tasks.md @@ -470,6 +470,8 @@ The following parameters apply to all task types. |`storeEmptyColumns`|Enables the task to store empty columns during ingestion. When `true`, Druid stores every column specified in the [`dimensionsSpec`](ingestion-spec.md#dimensionsspec). When `false`, Druid SQL queries referencing empty columns will fail. If you intend to leave `storeEmptyColumns` disabled, you should either ingest dummy data for empty columns or else not query on empty columns.

When set in the task context, `storeEmptyColumns` overrides the system property [`druid.indexer.task.storeEmptyColumns`](../configuration/index.md#additional-peon-configuration).|`true`| |`taskLockTimeout`|Task lock timeout in milliseconds. For more details, see [Locking](#locking).

When a task acquires a lock, it sends a request via HTTP and awaits until it receives a response containing the lock acquisition result. As a result, an HTTP timeout error can occur if `taskLockTimeout` is greater than `druid.server.http.maxIdleTime` of Overlords.|300000| |`useLineageBasedSegmentAllocation`|Enables the new lineage-based segment allocation protocol for the native Parallel task with dynamic partitioning. This option should be off during the replacing rolling upgrade from one of the Druid versions between 0.19 and 0.21 to Druid 0.22 or higher. Once the upgrade is done, it must be set to `true` to ensure data correctness.|`false` in 0.21 or earlier, `true` in 0.22 or later| +|`lookupLoadingMode`|Controls the lookup loading behavior in tasks. This property supports three values: `ALL` mode loads all the lookups, `NONE` mode does not load any lookups and `ONLY_REQUIRED` mode loads the lookups specified with context key `lookupsToLoad`. This property must not be specified for `MSQ` and `kill` tasks as the task engine enforces `ONLY_REQUIRED` mode for `MSQWorkerTask` and `NONE` mode for `MSQControllerTask` and `kill` tasks.|`ALL`| +|`lookupsToLoad`|List of lookup names to load in tasks. This property is required only if the `lookupLoadingMode` is set to `ONLY_REQUIRED`. For `MSQWorkerTask` type, the lookup names to load are identified by the controller task by parsing the SQL. |`null`| ## Task logs diff --git a/docs/multi-stage-query/reference.md b/docs/multi-stage-query/reference.md index 6facbaedcb3d..d34ba1bdd4e3 100644 --- a/docs/multi-stage-query/reference.md +++ b/docs/multi-stage-query/reference.md @@ -72,9 +72,8 @@ FROM TABLE( `name` and a `type`. The type can be `string`, `long`, `double`, or `float`. This row signature is used to map the external data into the SQL layer. -Variation 2, with the input schema expressed in SQL using an `EXTEND` clause. (See the next -section for more detail on `EXTEND`). This format also uses named arguments to make the -SQL a bit easier to read: +Variation 2, with the input schema expressed in SQL using an `EXTEND` clause. See the next +section for more detail on `EXTEND`. This format also uses named arguments to make the SQL easier to read: ```sql SELECT @@ -95,12 +94,12 @@ For more information, see [Read external data with EXTERN](concepts.md#read-exte #### `EXTERN` to export to a destination -`EXTERN` can be used to specify a destination where you want to export data to. -This variation of EXTERN requires one argument, the details of the destination as specified below. -This variation additionally requires an `AS` clause to specify the format of the exported rows. +You can use `EXTERN` to specify a destination to export data. +This variation of `EXTERN` accepts the details of the destination as the only argument and requires an `AS` clause to specify the format of the exported rows. + +When you export data, Druid creates metadata files in a subdirectory named `_symlink_format_manifest`. +Within the `_symlink_format_manifest/manifest` directory, the `manifest` file lists absolute paths to exported files using the symlink manifest format. For example: -While exporting data, some metadata files will also be created at the destination in addition to the data. These files will be created in a directory `_symlink_format_manifest`. -- `_symlink_format_manifest/manifest`: Lists the files which were created as part of the export. The file is in the symlink manifest format, and consists of a list of absolute paths to the files created. ```text s3://export-bucket/export/query-6564a32f-2194-423a-912e-eead470a37c4-worker2-partition2.csv s3://export-bucket/export/query-6564a32f-2194-423a-912e-eead470a37c4-worker1-partition1.csv @@ -112,8 +111,8 @@ s3://export-bucket/export/query-6564a32f-2194-423a-912e-eead470a37c4-worker0-par Keep the following in mind when using EXTERN to export rows: - Only INSERT statements are supported. - Only `CSV` format is supported as an export format. -- Partitioning (`PARTITIONED BY`) and clustering (`CLUSTERED BY`) aren't supported with export statements. -- You can export to Amazon S3 or local storage. +- Partitioning (`PARTITIONED BY`) and clustering (`CLUSTERED BY`) aren't supported with EXTERN statements. +- You can export to Amazon S3, Google GCS, or local storage. - The destination provided should contain no other files or directories. When you export data, use the `rowsPerPage` context parameter to restrict the size of exported files. @@ -128,10 +127,14 @@ SELECT FROM ``` -##### S3 +##### S3 - Amazon S3 + +To export results to S3, pass the `s3()` function as an argument to the `EXTERN` function. +Export to S3 requires the `druid-s3-extensions` extension. +For a list of S3 permissions the MSQ task engine requires to perform export, see [Permissions for durable storage](./security.md#s3). -Export results to S3 by passing the function `s3()` as an argument to the `EXTERN` function. Note that this requires the `druid-s3-extensions`. -The `s3()` function is a Druid function that configures the connection. Arguments for `s3()` should be passed as named parameters with the value in single quotes like the following example: +The `s3()` function configures the connection to AWS. +Pass all arguments for `s3()` as named parameters with their values enclosed in single quotes. For example: ```sql INSERT INTO @@ -146,25 +149,26 @@ FROM
Supported arguments for the function: -| Parameter | Required | Description | Default | -|-------------|----------|--------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------|---------| -| `bucket` | Yes | The S3 bucket to which the files are exported to. The bucket and prefix combination should be whitelisted in `druid.export.storage.s3.allowedExportPaths`. | n/a | -| `prefix` | Yes | Path where the exported files would be created. The export query expects the destination to be empty. If the location includes other files, then the query will fail. The bucket and prefix combination should be whitelisted in `druid.export.storage.s3.allowedExportPaths`. | n/a | +| Parameter | Required | Description | Default | +|---|---|---|---| +| `bucket` | Yes | S3 bucket destination for exported files. You must add the bucket and prefix combination to the `druid.export.storage.s3.allowedExportPaths` allow list. | n/a | +| `prefix` | Yes | Destination path in the bucket to create exported files. The export query expects the destination path to be empty. If the location includes other files, the query will fail. You must add the bucket and prefix combination to the `druid.export.storage.s3.allowedExportPaths` allow list. | n/a | -The following runtime parameters must be configured to export into an S3 destination: +Configure the following runtime parameters to export to an S3 destination: -| Runtime Parameter | Required | Description | Default | -|----------------------------------------------|----------|--------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------|-----| -| `druid.export.storage.s3.allowedExportPaths` | Yes | An array of S3 prefixes that are whitelisted as export destinations. Export queries fail if the export destination does not match any of the configured prefixes. Example: `[\"s3://bucket1/export/\", \"s3://bucket2/export/\"]` | n/a | -| `druid.export.storage.s3.tempLocalDir` | No | Directory used on the local storage of the worker to store temporary files required while uploading the data. Uses the task temporary directory by default. | n/a | -| `druid.export.storage.s3.maxRetry` | No | Defines the max number times to attempt S3 API calls to avoid failures due to transient errors. | 10 | -| `druid.export.storage.s3.chunkSize` | No | Defines the size of each chunk to temporarily store in `tempDir`. The chunk size must be between 5 MiB and 5 GiB. A large chunk size reduces the API calls to S3, however it requires more disk space to store the temporary chunks. | 100MiB | +| Runtime parameter | Required | Description | Default | +|---|---|---|---| +| `druid.export.storage.s3.allowedExportPaths` | Yes | Array of S3 prefixes allowed as export destinations. Export queries fail if the export destination does not match any of the configured prefixes. For example: `[\"s3://bucket1/export/\", \"s3://bucket2/export/\"]` | n/a | +| `druid.export.storage.s3.tempLocalDir` | No | Directory for local storage where the worker stores temporary files before uploading the data to S3. | n/a | +| `druid.export.storage.s3.maxRetry` | No | Maximum number of attempts for S3 API calls to avoid failures due to transient errors. | 10 | +| `druid.export.storage.s3.chunkSize` | No | Individual chunk size to store temporarily in `tempDir`. Large chunk sizes reduce the number of API calls to S3, but require more disk space to store temporary chunks. | 100MiB | +##### GOOGLE - Google Cloud Storage -##### GS +To export query results to Google Cloud Storage (GCS), pass the `google()` function as an argument to the `EXTERN` function. +Export to GCS requires the `druid-google-extensions` extension. -Export results to GCS by passing the function `google()` as an argument to the `EXTERN` function. Note that this requires the `druid-google-extensions`. -The `google()` function is a Druid function that configures the connection. Arguments for `google()` should be passed as named parameters with the value in single quotes like the following example: +The `google()` function configures the connection to GCS. Pass the arguments for `google()` as named parameters with their values enclosed in single quotes. For example: ```sql INSERT INTO @@ -179,29 +183,30 @@ FROM
Supported arguments for the function: -| Parameter | Required | Description | Default | -|-------------|----------|------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------|---------| -| `bucket` | Yes | The GS bucket to which the files are exported to. The bucket and prefix combination should be whitelisted in `druid.export.storage.google.allowedExportPaths`. | n/a | -| `prefix` | Yes | Path where the exported files would be created. The export query expects the destination to be empty. If the location includes other files, then the query will fail. The bucket and prefix combination should be whitelisted in `druid.export.storage.google.allowedExportPaths`. | n/a | +| Parameter | Required | Description | Default | +|---|---|---|---| +| `bucket` | Yes | GCS bucket destination for exported files. You must add the bucket and prefix combination to the `druid.export.storage.google.allowedExportPaths` allow list. | n/a | +| `prefix` | Yes | Destination path in the bucket to create exported files. The export query expects the destination path to be empty. If the location includes other files, the query will fail. You must add the bucket and prefix combination to the `druid.export.storage.google.allowedExportPaths` allow list. | n/a | -The following runtime parameters must be configured to export into a GCS destination: +Configure the following runtime parameters to export query results to a GCS destination: -| Runtime Parameter | Required | Description | Default | -|--------------------------------------------------|----------|-------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------|---------| -| `druid.export.storage.google.allowedExportPaths` | Yes | An array of GS prefixes that are allowed as export destinations. Export queries fail if the export destination does not match any of the configured prefixes. Example: `[\"gs://bucket1/export/\", \"gs://bucket2/export/\"]` | n/a | -| `druid.export.storage.google.tempLocalDir` | No | Directory used on the local storage of the worker to store temporary files required while uploading the data. Uses the task temporary directory by default. | n/a | -| `druid.export.storage.google.maxRetry` | No | Defines the max number times to attempt GS API calls to avoid failures due to transient errors. | 10 | -| `druid.export.storage.google.chunkSize` | No | Defines the size of each chunk to temporarily store in `tempDir`. A large chunk size reduces the API calls to GS; however, it requires more disk space to store the temporary chunks. | 4MiB | +| Runtime parameter | Required | Description | Default | +|---|---|---|---| +| `druid.export.storage.google.allowedExportPaths` | Yes | Array of GCS prefixes allowed as export destinations. Export queries fail if the export destination does not match any of the configured prefixes. For example: `[\"gs://bucket1/export/\", \"gs://bucket2/export/\"]` | n/a | +| `druid.export.storage.google.tempLocalDir` | No | Directory for local storage where the worker stores temporary files before uploading the data to GCS. | n/a | +| `druid.export.storage.google.maxRetry` | No | Maximum number of attempts for GCS API calls to avoid failures due to transient errors. | 10 | +| `druid.export.storage.google.chunkSize` | No | Individual chunk size to store temporarily in `tempDir`. Large chunk sizes reduce the number of API calls to GS, but require more disk space to store temporary chunks. | 4 MiB | -##### LOCAL -You can export to the local storage, which exports the results to the filesystem of the MSQ worker. +##### LOCAL - local file storage + +You can export queries to local storage. This process writes the results to the filesystem on the MSQ worker. This is useful in a single node setup or for testing but is not suitable for production use cases. -Export results to local storage by passing the function `LOCAL()` as an argument for the `EXTERN FUNCTION`. -To use local storage as an export destination, the runtime property `druid.export.storage.baseDir` must be configured on the Indexer/Middle Manager. -This value must be set to an absolute path on the local machine. Exporting data will be allowed to paths which match the prefix set by this value. -Arguments to `LOCAL()` should be passed as named parameters with the value in single quotes in the following example: +To export results to local storage, pass the `LOCAL()` function as an argument to the EXTERN function. +You must configure the runtime property `druid.export.storage.baseDir` as an absolute path on the Indexer or Middle Manager to use local storage as an export destination. +You can export data to paths that match this value as a prefix. +Pass all arguments to `LOCAL()` as named parameters with values enclosed in single quotes. For example: ```sql INSERT INTO @@ -214,13 +219,13 @@ SELECT FROM
``` -Supported arguments to the function: +Supported arguments for the function: -| Parameter | Required | Description | Default | -|-------------|--------------------------------|----------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------| --| -| `exportPath` | Yes | Absolute path to a subdirectory of `druid.export.storage.baseDir` used as the destination to export the results to. The export query expects the destination to be empty. If the location includes other files or directories, then the query will fail. | n/a | +| Parameter | Required | Description | Default | +|---|---|---|---| +| `exportPath` | Yes | Absolute path to a subdirectory of `druid.export.storage.baseDir` where Druid exports the query results. The destination must be empty. If the location includes other files or directories, the query will fail. | n/a | -For more information, see [Read external data with EXTERN](concepts.md#write-to-an-external-destination-with-extern). +For more information, see [Export external data with EXTERN](concepts.md#write-to-an-external-destination-with-extern). ### `INSERT` diff --git a/docs/multi-stage-query/security.md b/docs/multi-stage-query/security.md index 2aed00ab851f..d98695ebedce 100644 --- a/docs/multi-stage-query/security.md +++ b/docs/multi-stage-query/security.md @@ -81,3 +81,6 @@ The MSQ task engine needs the following permissions for pushing, fetching, and r - `Microsoft.Storage/storageAccounts/blobServices/containers/blobs/write` to write files in durable storage. - `Microsoft.Storage/storageAccounts/blobServices/containers/blobs/add/action` to create files in durable storage. - `Microsoft.Storage/storageAccounts/blobServices/containers/blobs/delete` to delete files when they're no longer needed. + + + diff --git a/docs/operations/metrics.md b/docs/operations/metrics.md index ccb20f2e6dd1..7258e965a11c 100644 --- a/docs/operations/metrics.md +++ b/docs/operations/metrics.md @@ -177,9 +177,9 @@ If SQL is enabled, the Broker will emit the following metrics for SQL. |Metric|Description|Dimensions|Normal value| |------|-----------|----------|------------| -|`sqlQuery/time`|Milliseconds taken to complete a SQL.|`id`, `nativeQueryIds`, `dataSource`, `remoteAddress`, `success`|< 1s| -|`sqlQuery/planningTimeMs`|Milliseconds taken to plan a SQL to native query.|`id`, `nativeQueryIds`, `dataSource`, `remoteAddress`, `success`| | -|`sqlQuery/bytes`|number of bytes returned in SQL response.|`id`, `nativeQueryIds`, `dataSource`, `remoteAddress`, `success`| | +|`sqlQuery/time`|Milliseconds taken to complete a SQL.|`id`, `nativeQueryIds`, `dataSource`, `remoteAddress`, `success`, `engine`|< 1s| +|`sqlQuery/planningTimeMs`|Milliseconds taken to plan a SQL to native query.|`id`, `nativeQueryIds`, `dataSource`, `remoteAddress`, `success`, `engine`| | +|`sqlQuery/bytes`|number of bytes returned in SQL response.|`id`, `nativeQueryIds`, `dataSource`, `remoteAddress`, `success`, `engine`| | ## Ingestion metrics @@ -402,7 +402,7 @@ These metrics are emitted by the Druid Coordinator in every run of the correspon |Metric|Description|Dimensions|Normal value| |------|-----------|----------|------------| -| `service/heartbeat` | Metric indicating the service is up. This metric is emitted only when `ServiceStatusMonitor` is enabled. | `leader` on the Overlord and Coordinator.
`workerVersion`, `category`, `status` on the Middle Manager.
`taskId`, `groupId`, `taskType`, `dataSource`, `tags` on the Peon |1| +| `service/heartbeat` | Metric indicating the service is up. This metric is emitted only when `ServiceStatusMonitor` is enabled. | `leader` on the Overlord and Coordinator.
`workerVersion`, `category`, `status` on the Middle Manager.
`taskId`, `groupId`, `taskType`, `status`, `dataSource`, `tags` on the Peon |1| ### Historical diff --git a/docs/querying/filters.md b/docs/querying/filters.md index 20b270903151..ae84ecbfbafb 100644 --- a/docs/querying/filters.md +++ b/docs/querying/filters.md @@ -23,6 +23,11 @@ sidebar_label: "Filters" ~ under the License. --> +A filter is a JSON object indicating which rows of data should be included in the computation for a query. It’s essentially the equivalent of the WHERE clause in SQL. +Filters are commonly applied on dimensions, but can be applied on aggregated metrics, for example, see [Filtered aggregator](./aggregations.md#filtered-aggregator) and [Having filters](./having.md). + +By default, Druid uses SQL compatible three-value logic when filtering. See [Boolean logic](./sql-data-types.md#boolean-logic) for more details. + :::info Apache Druid supports two query languages: [Druid SQL](sql.md) and [native queries](querying.md). This document describes the native @@ -30,12 +35,7 @@ sidebar_label: "Filters" [SQL documentation](sql-scalar.md). ::: -A filter is a JSON object indicating which rows of data should be included in the computation for a query. It’s essentially the equivalent of the WHERE clause in SQL. -Filters are commonly applied on dimensions, but can be applied on aggregated metrics, for example, see [Filtered aggregator](./aggregations.md#filtered-aggregator) and [Having filters](./having.md). - -By default, Druid uses SQL compatible three-value logic when filtering. See [Boolean logic](./sql-data-types.md#boolean-logic) for more details. - -Apache Druid supports the following types of filters. +This topic describes the filter types supported in Apache Druid. ## Selector filter @@ -43,7 +43,7 @@ The simplest filter is a selector filter. The selector filter matches a specific | Property | Description | Required | | -------- | ----------- | -------- | -| `type` | Must be "selector".| Yes | +| `type` | Must be `selector`.| Yes | | `dimension` | Input column or virtual column name to filter on. | Yes | | `value` | String value to match. | No. If not specified the filter matches NULL values. | | `extractionFn` | [Extraction function](./dimensionspecs.md#extraction-functions) to apply to `dimension` prior to value matching. See [filtering with extraction functions](#filtering-with-extraction-functions) for details. | No | @@ -52,21 +52,19 @@ The selector filter can only match against `STRING` (single and multi-valued), ` When the selector filter matches against numeric inputs, the string `value` will be best-effort coerced into a numeric value. -### Example: equivalent of `WHERE someColumn = 'hello'` +**Example**: equivalent of `WHERE someColumn = 'hello'` ``` json { "type": "selector", "dimension": "someColumn", "value": "hello" } ``` - -### Example: equivalent of `WHERE someColumn IS NULL` +**Example**: equivalent of `WHERE someColumn IS NULL` ``` json { "type": "selector", "dimension": "someColumn", "value": null } ``` - -## Equality Filter +## Equality filter The equality filter is a replacement for the selector filter with the ability to match against any type of column. The equality filter is designed to have more SQL compatible behavior than the selector filter and so can not match null values. To match null values use the null filter. @@ -74,31 +72,30 @@ Druid's SQL planner uses the equality filter by default instead of selector filt | Property | Description | Required | | -------- | ----------- | -------- | -| `type` | Must be "equality".| Yes | +| `type` | Must be `equals`.| Yes | | `column` | Input column or virtual column name to filter on. | Yes | | `matchValueType` | String specifying the type of value to match. For example `STRING`, `LONG`, `DOUBLE`, `FLOAT`, `ARRAY`, `ARRAY`, or any other Druid type. The `matchValueType` determines how Druid interprets the `matchValue` to assist in converting to the type of the matched `column`. | Yes | | `matchValue` | Value to match, must not be null. | Yes | -### Example: equivalent of `WHERE someColumn = 'hello'` +**Example**: equivalent of `WHERE someColumn = 'hello'` ```json { "type": "equals", "column": "someColumn", "matchValueType": "STRING", "matchValue": "hello" } ``` -### Example: equivalent of `WHERE someNumericColumn = 1.23` +**Example**: equivalent of `WHERE someNumericColumn = 1.23` ```json { "type": "equals", "column": "someNumericColumn", "matchValueType": "DOUBLE", "matchValue": 1.23 } ``` -### Example: equivalent of `WHERE someArrayColumn = ARRAY[1, 2, 3]` +**Example**: equivalent of `WHERE someArrayColumn = ARRAY[1, 2, 3]` ```json { "type": "equals", "column": "someArrayColumn", "matchValueType": "ARRAY", "matchValue": [1, 2, 3] } ``` - -## Null Filter +## Null filter The null filter is a partial replacement for the selector filter. It is dedicated to matching NULL values. @@ -106,30 +103,29 @@ Druid's SQL planner uses the null filter by default instead of selector filter w | Property | Description | Required | | -------- | ----------- | -------- | -| `type` | Must be "null".| Yes | +| `type` | Must be `null`.| Yes | | `column` | Input column or virtual column name to filter on. | Yes | -### Example: equivalent of `WHERE someColumn IS NULL` +**Example**: equivalent of `WHERE someColumn IS NULL` ```json { "type": "null", "column": "someColumn" } ``` - ## Column comparison filter The column comparison filter is similar to the selector filter, but compares dimensions to each other. For example: | Property | Description | Required | | -------- | ----------- | -------- | -| `type` | Must be "selector".| Yes | +| `type` | Must be `selector`.| Yes | | `dimensions` | List of [`DimensionSpec`](./dimensionspecs.md) to compare. | Yes | `dimensions` is list of [DimensionSpecs](./dimensionspecs.md), making it possible to apply an extraction function if needed. Note that the column comparison filter converts all values to strings prior to comparison. This allows differently-typed input columns to match without a cast operation. -### Example: equivalent of `WHERE someColumn = someLongColumn` +**Example**: equivalent of `WHERE someColumn = someLongColumn` ``` json { @@ -145,18 +141,17 @@ Note that the column comparison filter converts all values to strings prior to c } ``` - ## Logical expression filters ### AND | Property | Description | Required | | -------- | ----------- | -------- | -| `type` | Must be "and".| Yes | +| `type` | Must be `and`.| Yes | | `fields` | List of filter JSON objects, such as any other filter defined on this page or provided by extensions. | Yes | -#### Example: equivalent of `WHERE someColumn = 'a' AND otherColumn = 1234 AND anotherColumn IS NULL` +**Example**: equivalent of `WHERE someColumn = 'a' AND otherColumn = 1234 AND anotherColumn IS NULL` ``` json { @@ -173,10 +168,10 @@ Note that the column comparison filter converts all values to strings prior to c | Property | Description | Required | | -------- | ----------- | -------- | -| `type` | Must be "or".| Yes | +| `type` | Must be `or`.| Yes | | `fields` | List of filter JSON objects, such as any other filter defined on this page or provided by extensions. | Yes | -#### Example: equivalent of `WHERE someColumn = 'a' OR otherColumn = 1234 OR anotherColumn IS NULL` +**Example**: equivalent of `WHERE someColumn = 'a' OR otherColumn = 1234 OR anotherColumn IS NULL` ``` json { @@ -193,22 +188,21 @@ Note that the column comparison filter converts all values to strings prior to c | Property | Description | Required | | -------- | ----------- | -------- | -| `type` | Must be "not".| Yes | +| `type` | Must be `not`.| Yes | | `field` | Filter JSON objects, such as any other filter defined on this page or provided by extensions. | Yes | -#### Example: equivalent of `WHERE someColumn IS NOT NULL` +**Example**: equivalent of `WHERE someColumn IS NOT NULL` ```json { "type": "not", "field": { "type": "null", "column": "someColumn" }} ``` - ## In filter The in filter can match input rows against a set of values, where a match occurs if the value is contained in the set. | Property | Description | Required | | -------- | ----------- | -------- | -| `type` | Must be "in".| Yes | +| `type` | Must be `in`.| Yes | | `dimension` | Input column or virtual column name to filter on. | Yes | | `values` | List of string value to match. | Yes | | `extractionFn` | [Extraction function](./dimensionspecs.md#extraction-functions) to apply to `dimension` prior to value matching. See [filtering with extraction functions](#filtering-with-extraction-functions) for details. | No | @@ -219,7 +213,7 @@ If an empty `values` array is passed to the "in" filter, it will simply return a If the `values` array contains `null`, the "in" filter matches null values. This differs from the SQL IN filter, which does not match NULL values. -### Example: equivalent of `WHERE `outlaw` IN ('Good', 'Bad', 'Ugly')` +**Example**: equivalent of `WHERE `outlaw` IN ('Good', 'Bad', 'Ugly')` ```json { @@ -229,7 +223,6 @@ does not match NULL values. } ``` - ## Bound filter Bound filters can be used to filter on ranges of dimension values. It can be used for comparison filtering like @@ -238,7 +231,7 @@ greater than, less than, greater than or equal to, less than or equal to, and "b | Property | Description | Required | | -------- | ----------- | -------- | -| `type` | Must be "bound". | Yes | +| `type` | Must be `bound`. | Yes | | `dimension` | Input column or virtual column name to filter on. | Yes | | `lower` | The lower bound string match value for the filter. | No | | `upper`| The upper bound string match value for the filter. | No | @@ -253,7 +246,7 @@ The bound filter can only match against `STRING` (single and multi-valued), `LON Note that the bound filter matches null values if you don't specify a lower bound. Use the range filter if SQL-compatible behavior. -### Example: equivalent to `WHERE 21 <= age <= 31` +**Example**: equivalent to `WHERE 21 <= age <= 31` ```json { @@ -265,7 +258,7 @@ Note that the bound filter matches null values if you don't specify a lower boun } ``` -### Example: equivalent to `WHERE 'foo' <= name <= 'hoo'`, using the default lexicographic sorting order +**Example**: equivalent to `WHERE 'foo' <= name <= 'hoo'`, using the default lexicographic sorting order ```json { @@ -276,7 +269,7 @@ Note that the bound filter matches null values if you don't specify a lower boun } ``` -### Example: equivalent to `WHERE 21 < age < 31` +**Example**: equivalent to `WHERE 21 < age < 31` ```json { @@ -290,7 +283,7 @@ Note that the bound filter matches null values if you don't specify a lower boun } ``` -### Example: equivalent to `WHERE age < 31` +**Example**: equivalent to `WHERE age < 31` ```json { @@ -302,7 +295,7 @@ Note that the bound filter matches null values if you don't specify a lower boun } ``` -### Example: equivalent to `WHERE age >= 18` +**Example**: equivalent to `WHERE age >= 18` ```json { @@ -313,7 +306,6 @@ Note that the bound filter matches null values if you don't specify a lower boun } ``` - ## Range filter The range filter is a replacement for the bound filter. It compares against any type of column and is designed to have has more SQL compliant behavior than the bound filter. It won't match null values, even if you don't specify a lower bound. @@ -322,7 +314,7 @@ Druid's SQL planner uses the range filter by default instead of bound filter whe | Property | Description | Required | | -------- | ----------- | -------- | -| `type` | Must be "range".| Yes | +| `type` | Must be `range`.| Yes | | `column` | Input column or virtual column name to filter on. | Yes | | `matchValueType` | String specifying the type of bounds to match. For example `STRING`, `LONG`, `DOUBLE`, `FLOAT`, `ARRAY`, `ARRAY`, or any other Druid type. The `matchValueType` determines how Druid interprets the `matchValue` to assist in converting to the type of the matched `column` and also defines the type of comparison used when matching values. | Yes | | `lower` | Lower bound value to match. | No. At least one of `lower` or `upper` must not be null. | @@ -330,7 +322,7 @@ Druid's SQL planner uses the range filter by default instead of bound filter whe | `lowerOpen` | Boolean indicating if lower bound is open in the interval of values defined by the range (">" instead of ">="). | No | | `upperOpen` | Boolean indicating if upper bound is open on the interval of values defined by range ("<" instead of "<="). | No | -### Example: equivalent to `WHERE 21 <= age <= 31` +**Example**: equivalent to `WHERE 21 <= age <= 31` ```json { @@ -342,7 +334,7 @@ Druid's SQL planner uses the range filter by default instead of bound filter whe } ``` -### Example: equivalent to `WHERE 'foo' <= name <= 'hoo'`, using STRING comparison +**Example**: equivalent to `WHERE 'foo' <= name <= 'hoo'`, using STRING comparison ```json { @@ -354,7 +346,7 @@ Druid's SQL planner uses the range filter by default instead of bound filter whe } ``` -### Example: equivalent to `WHERE 21 < age < 31` +**Example**: equivalent to `WHERE 21 < age < 31` ```json { @@ -368,7 +360,7 @@ Druid's SQL planner uses the range filter by default instead of bound filter whe } ``` -### Example: equivalent to `WHERE age < 31` +**Example**: equivalent to `WHERE age < 31` ```json { @@ -380,7 +372,7 @@ Druid's SQL planner uses the range filter by default instead of bound filter whe } ``` -### Example: equivalent to `WHERE age >= 18` +**Example**: equivalent to `WHERE age >= 18` ```json { @@ -391,7 +383,7 @@ Druid's SQL planner uses the range filter by default instead of bound filter whe } ``` -### Example: equivalent to `WHERE ARRAY['a','b','c'] < arrayColumn < ARRAY['d','e','f']`, using ARRAY comparison +**Example**: equivalent to `WHERE ARRAY['a','b','c'] < arrayColumn < ARRAY['d','e','f']`, using ARRAY comparison ```json { @@ -413,7 +405,7 @@ supported are "%" (matches any number of characters) and "\_" (matches any one c | Property | Description | Required | | -------- | ----------- | -------- | -| `type` | Must be "like".| Yes | +| `type` | Must be `like`.| Yes | | `dimension` | Input column or virtual column name to filter on. | Yes | | `pattern` | String LIKE pattern, such as "foo%" or "___bar".| Yes | | `escape`| A string escape character that can be used to escape special characters. | No | @@ -421,7 +413,7 @@ supported are "%" (matches any number of characters) and "\_" (matches any one c Like filters support the use of extraction functions, see [Filtering with Extraction Functions](#filtering-with-extraction-functions) for details. -### Example: equivalent of `WHERE last_name LIKE "D%"` (last_name starts with "D") +**Example**: equivalent of `WHERE last_name LIKE "D%"` (last_name starts with "D") ```json { @@ -437,14 +429,14 @@ The regular expression filter is similar to the selector filter, but using regul | Property | Description | Required | | -------- | ----------- | -------- | -| `type` | Must be "regex".| Yes | +| `type` | Must be `regex`.| Yes | | `dimension` | Input column or virtual column name to filter on. | Yes | | `pattern` | String pattern to match - any standard [Java regular expression](https://docs.oracle.com/en/java/javase/11/docs/api/java.base/java/util/regex/Pattern.html). | Yes | | `extractionFn` | [Extraction function](./dimensionspecs.md#extraction-functions) to apply to `dimension` prior to value matching. See [filtering with extraction functions](#filtering-with-extraction-functions) for details. | No | Note that it is often more optimal to use a like filter instead of a regex for simple matching of prefixes. -### Example: matches values that start with "50." +**Example**: matches values that start with `50.` ``` json { "type": "regex", "dimension": "someColumn", "pattern": ^50.* } @@ -456,24 +448,24 @@ The `arrayContainsElement` filter checks if an `ARRAY` contains a specific eleme | Property | Description | Required | | -------- | ----------- | -------- | -| `type` | Must be "arrayContainsElement".| Yes | +| `type` | Must be `arrayContainsElement`.| Yes | | `column` | Input column or virtual column name to filter on. | Yes | | `elementMatchValueType` | String specifying the type of element value to match. For example `STRING`, `LONG`, `DOUBLE`, `FLOAT`, `ARRAY`, `ARRAY`, or any other Druid type. The `elementMatchValueType` determines how Druid interprets the `elementMatchValue` to assist in converting to the type of elements contained in the matched `column`. | Yes | | `elementMatchValue` | Array element value to match. This value can be null. | Yes | -### Example: equivalent of `WHERE ARRAY_CONTAINS(someArrayColumn, 'hello')` +**Example**: equivalent of `WHERE ARRAY_CONTAINS(someArrayColumn, 'hello')` ```json { "type": "arrayContainsElement", "column": "someArrayColumn", "elementMatchValueType": "STRING", "elementMatchValue": "hello" } ``` -### Example: equivalent of `WHERE ARRAY_CONTAINS(someNumericArrayColumn, 1.23)` +**Example**: equivalent of `WHERE ARRAY_CONTAINS(someNumericArrayColumn, 1.23)` ```json { "type": "arrayContainsElement", "column": "someNumericArrayColumn", "elementMatchValueType": "DOUBLE", "elementMatchValue": 1.23 } ``` -### Example: equivalent of `WHERE ARRAY_CONTAINS(someNumericArrayColumn, ARRAY[1, 2, 3])` +**Example**: equivalent of `WHERE ARRAY_CONTAINS(someNumericArrayColumn, ARRAY[1, 2, 3])` ```json { @@ -487,7 +479,7 @@ The `arrayContainsElement` filter checks if an `ARRAY` contains a specific eleme ``` -### Example: equivalent of `WHERE ARRAY_OVERLAPS(someNumericArrayColumn, ARRAY[1, 2, 3])` +**Example**: equivalent of `WHERE ARRAY_OVERLAPS(someNumericArrayColumn, ARRAY[1, 2, 3])` ```json { @@ -508,7 +500,7 @@ This filter converts the ISO 8601 intervals to long millisecond start/end ranges | Property | Description | Required | | -------- | ----------- | -------- | -| `type` | Must be "interval". | Yes | +| `type` | Must be `interval`. | Yes | | `dimension` | Input column or virtual column name to filter on. | Yes | | `intervals` | A JSON array containing ISO-8601 interval strings that defines the time ranges to filter on. | Yes | | `extractionFn` | [Extraction function](./dimensionspecs.md#extraction-functions) to apply to `dimension` prior to value matching. See [filtering with extraction functions](#filtering-with-extraction-functions) for details. | No | @@ -558,8 +550,8 @@ The filter above is equivalent to the following OR of Bound filters: } ``` - ## True filter + A filter which matches all values. You can use it to temporarily disable other filters without removing them. ```json @@ -567,13 +559,13 @@ A filter which matches all values. You can use it to temporarily disable other f ``` ## False filter + A filter matches no values. You can use it to force a query to match no values. ```json {"type": "false" } ``` - ## Search filter You can use search filters to filter on partial string matches. @@ -593,7 +585,7 @@ You can use search filters to filter on partial string matches. | Property | Description | Required | | -------- | ----------- | -------- | -| `type` | Must be "search". | Yes | +| `type` | Must be `search`. | Yes | | `dimension` | Input column or virtual column name to filter on. | Yes | | `query`| A JSON object for the type of search. See [search query spec](#search-query-spec) for more information. | Yes | | `extractionFn` | [Extraction function](./dimensionspecs.md#extraction-functions) to apply to `dimension` prior to value matching. See [filtering with extraction functions](#filtering-with-extraction-functions) for details. | No | @@ -604,7 +596,7 @@ You can use search filters to filter on partial string matches. | Property | Description | Required | | -------- | ----------- | -------- | -| `type` | Must be "contains". | Yes | +| `type` | Must be `contains`. | Yes | | `value` | A String value to search. | Yes | | `caseSensitive` | Whether the string comparison is case-sensitive or not. | No, default is false (insensitive) | @@ -612,7 +604,7 @@ You can use search filters to filter on partial string matches. | Property | Description | Required | | -------- | ----------- | -------- | -| `type` | Must be "insensitive_contains". | Yes | +| `type` | Must be `insensitive_contains`. | Yes | | `value` | A String value to search. | Yes | Note that an "insensitive_contains" search is equivalent to a "contains" search with "caseSensitive": false (or not @@ -622,22 +614,20 @@ provided). | Property | Description | Required | | -------- | ----------- | -------- | -| `type` | Must be "fragment". | Yes | +| `type` | Must be `fragment`. | Yes | | `values` | A JSON array of string values to search. | Yes | | `caseSensitive` | Whether the string comparison is case-sensitive or not. | No, default is false (insensitive) | - - ## Expression filter The expression filter allows for the implementation of arbitrary conditions, leveraging the Druid expression system. This filter allows for complete flexibility, but it might be less performant than a combination of the other filters on this page because it can't always use the same optimizations available to other filters. | Property | Description | Required | | -------- | ----------- | -------- | -| `type` | Must be "expression" | Yes | +| `type` | Must be `expression`. | Yes | | `expression` | Expression string to evaluate into true or false. See the [Druid expression system](math-expr.md) for more details. | Yes | -### Example: expression based matching +**Example**: expression based matching ```json { @@ -646,19 +636,18 @@ The expression filter allows for the implementation of arbitrary conditions, lev } ``` - ## JavaScript filter The JavaScript filter matches a dimension against the specified JavaScript function predicate. The filter matches values for which the function returns true. | Property | Description | Required | | -------- | ----------- | -------- | -| `type` | Must be "javascript" | Yes | +| `type` | Must be `javascript`. | Yes | | `dimension` | Input column or virtual column name to filter on. | Yes | | `function` | JavaScript function which accepts the dimension value as a single argument, and returns either true or false. | Yes | | `extractionFn` | [Extraction function](./dimensionspecs.md#extraction-functions) to apply to `dimension` prior to value matching. See [filtering with extraction functions](#filtering-with-extraction-functions) for details. | No | -### Example: matching any dimension values for the dimension `name` between `'bar'` and `'foo'` +**Example**: matching any dimension values for the dimension `name` between `'bar'` and `'foo'` ```json { @@ -685,12 +674,12 @@ The following filter matches the values for which the extraction function has a | Property | Description | Required | | -------- | ----------- | -------- | -| `type` | Must be "extraction" | Yes | +| `type` | Must be `extraction`. | Yes | | `dimension` | Input column or virtual column name to filter on. | Yes | | `value` | String value to match. | No. If not specified the filter will match NULL values. | | `extractionFn` | [Extraction function](./dimensionspecs.md#extraction-functions) to apply to `dimension` prior to value matching. See [filtering with extraction functions](#filtering-with-extraction-functions) for details. | No | -### Example: matching dimension values in `[product_1, product_3, product_5]` for the column `product` +**Example**: matching dimension values in `[product_1, product_3, product_5]` for the column `product` ```json { @@ -723,7 +712,7 @@ If specified, the extraction function will be used to transform input values bef The example below shows a selector filter combined with an extraction function. This filter will transform input values according to the values defined in the lookup map; transformed values will then be matched with the string "bar_1". -### Example: matches dimension values in `[product_1, product_3, product_5]` for the column `product` +**Example**: matches dimension values in `[product_1, product_3, product_5]` for the column `product` ```json { @@ -757,7 +746,8 @@ scan those columns. All filters return true if any one of the dimension values is satisfies the filter. -#### Example: multi-value match behavior +**Example**: multi-value match behavior + Given a multi-value STRING row with values `['a', 'b', 'c']`, a filter such as ```json @@ -767,7 +757,8 @@ will successfully match the entire row. This can produce sometimes unintuitive b Additionally, contradictory filters may be defined and perfectly legal in native queries which will not work in SQL. -#### Example: SQL "contradiction" +**Example**: SQL "contradiction" + This query is impossible to express as is in SQL since it is a contradiction that the SQL planner will optimize to false and match nothing. Given a multi-value STRING row with values `['a', 'b', 'c']`, and filter such as @@ -802,7 +793,7 @@ When filtering on numeric columns using string based filters such as the selecto converted into a numeric predicate and will be applied to the numeric column values directly. In some cases (such as the "regex" filter) the numeric column values will be converted to strings during the scan. -#### Example: filtering on a specific value, `myFloatColumn = 10.1` +**Example**: filtering on a specific value, `myFloatColumn = 10.1` ```json { @@ -823,7 +814,7 @@ or with a selector filter: } ``` -#### Example: filtering on a range of values, `10 <= myFloatColumn < 20` +**Example**: filtering on a range of values, `10 <= myFloatColumn < 20` ```json { @@ -859,7 +850,7 @@ should be specified as if the timestamp values were strings. If you want to interpret the timestamp with a specific format, timezone, or locale, the [Time Format Extraction Function](./dimensionspecs.md#time-format-extraction-function) is useful. -#### Example: filtering on a long timestamp value +**Example**: filtering on a long timestamp value ```json { @@ -880,7 +871,7 @@ or with a selector filter: } ``` -#### Example: filtering on day of week using an extraction function +**Example**: filtering on day of week using an extraction function ```json { @@ -896,7 +887,7 @@ or with a selector filter: } ``` -#### Example: filtering on a set of ISO 8601 intervals +**Example**: filtering on a set of ISO 8601 intervals ```json { @@ -908,4 +899,3 @@ or with a selector filter: ] } ``` - diff --git a/docs/querying/topnquery.md b/docs/querying/topnquery.md index 663ae2ce7db5..25d8b2248028 100644 --- a/docs/querying/topnquery.md +++ b/docs/querying/topnquery.md @@ -32,7 +32,7 @@ sidebar_label: "TopN" Apache Druid TopN queries return a sorted set of results for the values in a given dimension according to some criteria. Conceptually, they can be thought of as an approximate [GroupByQuery](../querying/groupbyquery.md) over a single dimension with an [Ordering](../querying/limitspec.md) spec. TopNs are much faster and resource efficient than GroupBys for this use case. These types of queries take a topN query object and return an array of JSON objects where each object represents a value asked for by the topN query. -TopNs are approximate in that each data process will rank their top K results and only return those top K results to the Broker. K, by default in Druid, is `max(1000, threshold)`. In practice, this means that if you ask for the top 1000 items ordered, the correctness of the first ~900 items will be 100%, and the ordering of the results after that is not guaranteed. TopNs can be made more accurate by increasing the threshold. +TopNs are approximate in that each data process will rank their top K results and only return those top K results to the Broker. K, by default in Druid, is `max(1000, threshold)`. A topN query object looks like: diff --git a/docs/tutorials/tutorial-extern.md b/docs/tutorials/tutorial-extern.md new file mode 100644 index 000000000000..d44dd19a1542 --- /dev/null +++ b/docs/tutorials/tutorial-extern.md @@ -0,0 +1,206 @@ +--- +id: tutorial-extern +title: Export query results +sidebar_label: Export results +description: How to use EXTERN to export query results. +--- + + + +import Tabs from '@theme/Tabs'; +import TabItem from '@theme/TabItem'; + +This tutorial demonstrates how to use the Apache Druid® SQL [EXTERN](../multi-stage-query/reference.md#extern-function) function to export data. + +## Prerequisites + +Before you follow the steps in this tutorial, download Druid as described in the [Local quickstart](index.md). +Don't start Druid, you'll do that as part of the tutorial. + +You should be familiar with ingesting and querying data in Druid. +If you haven't already, go through the [Query data](../tutorials/tutorial-query.md) tutorial first. + +## Export query results to the local file system + +This example demonstrates how to configure Druid to export data to the local file system. +While you can use this approach to learn about EXTERN syntax for exporting data, it's not suitable for production scenarios. + +### Configure Druid local export directory + +The following commands set the base path for the Druid exports to `/tmp/druid/`. +If the account running Druid doesn't have access to `/tmp/druid/`, change the path. +For example: `/Users/Example/druid`. +If you change the path in this step, use the updated path in all subsequent steps. + +From the root of the Druid distribution, run the following: + +```bash +export export_path="/tmp/druid" +sed -i -e $'$a\\\n\\\n\\\n#\\\n###Local export\\\n#\\\ndruid.export.storage.baseDir='$export_path' conf/druid/auto/_common/common.runtime.properties +``` + +This adds the following section to the Druid `common.runtime.properties` configuration file located in `conf/druid/auto/_common`: + +``` +# +###Local export +# +druid.export.storage.baseDir=/tmp/druid/ +``` + +### Start Druid and load sample data + +1. From the root of the Druid distribution, launch Druid as follows: + + ```bash + ./bin/start-druid + ``` +1. After Druid starts, open [http://localhost:8888/](http://localhost:8888/) in your browser to access the Web Console. +1. From the [Query view](http://localhost:8888/unified-console.html#workbench), run the following command to load the Wikipedia example data set: + ```sql + REPLACE INTO "wikipedia" OVERWRITE ALL + WITH "ext" AS ( + SELECT * + FROM TABLE( + EXTERN( + '{"type":"http","uris":["https://druid.apache.org/data/wikipedia.json.gz"]}', + '{"type":"json"}' + ) + ) EXTEND ("isRobot" VARCHAR, "channel" VARCHAR, "timestamp" VARCHAR, "flags" VARCHAR, "isUnpatrolled" VARCHAR, "page" VARCHAR, "diffUrl" VARCHAR, "added" BIGINT, "comment" VARCHAR, "commentLength" BIGINT, "isNew" VARCHAR, "isMinor" VARCHAR, "delta" BIGINT, "isAnonymous" VARCHAR, "user" VARCHAR, "deltaBucket" BIGINT, "deleted" BIGINT, "namespace" VARCHAR, "cityName" VARCHAR, "countryName" VARCHAR, "regionIsoCode" VARCHAR, "metroCode" BIGINT, "countryIsoCode" VARCHAR, "regionName" VARCHAR) + ) + SELECT + TIME_PARSE("timestamp") AS "__time", + "isRobot", + "channel", + "flags", + "isUnpatrolled", + "page", + "diffUrl", + "added", + "comment", + "commentLength", + "isNew", + "isMinor", + "delta", + "isAnonymous", + "user", + "deltaBucket", + "deleted", + "namespace", + "cityName", + "countryName", + "regionIsoCode", + "metroCode", + "countryIsoCode", + "regionName" + FROM "ext" + PARTITIONED BY DAY + ``` + +### Query to export data + +Open a new tab and run the following query to export query results to the path: +`/tmp/druid/wiki_example`. +The path must be a subdirectory of the `druid.export.storage.baseDir`. + + +```sql +INSERT INTO + EXTERN( + local(exportPath => '/tmp/druid/wiki_example') + ) +AS CSV +SELECT "channel", + SUM("delta") AS "changes" +FROM "wikipedia" +GROUP BY 1 +LIMIT 10 +``` + +Druid exports the results of the query to the `/tmp/druid/wiki_example` directory. +Run the following command to list the contents of + +```bash +ls /tmp/druid/wiki_example +``` + +The results are a CSV file export of the data and a directory. + +## Export query results to cloud storage + +The steps to export to cloud storage are similar to exporting to the local file system. +Druid supports Amazon S3 or Google Cloud Storage (GCS) as cloud storage destinations. + +1. Enable the extension for your cloud storage destination. See [Loading core extensions](../configuration/extensions.md#loading-core-extensions). + - **Amazon S3**: `druid-s3-extensions` + - **GCS**: `google-extensions` + See [Loading core extensions](../configuration/extensions.md#loading-core-extensions) for more information. +1. Configure the additional properties for your cloud storage destination. Replace `{CLOUD}` with `s3` or `google` accordingly: + - `druid.export.storage.{CLOUD}.tempLocalDir`: Local temporary directory where the query engine stages files to export. + - `druid.export.storage.{CLOUD}.allowedExportPaths`: S3 or GS prefixes allowed as Druid export locations. For example `[\"s3://bucket1/export/\",\"s3://bucket2/export/\"]` or `[\"gs://bucket1/export/\", \"gs://bucket2/export/\"]`. + - `druid.export.storage.{CLOUD}.maxRetry`: Maximum number of times to attempt cloud API calls to avoid failures from transient errors. + - `druid.export.storage.s3.chunkSize`: Maximum size of individual data chunks to store in the temporary directory. +1. Verify the instance role has the correct permissions to the bucket and folders: read, write, create, and delete. See [Permissions for durable storage](../multi-stage-query/security.md#permissions-for-durable-storage). +1. Use the query syntax for your cloud storage type. For example: + + + + + + ```sql + INSERT INTO + EXTERN( + s3(bucket => 'your_bucket', prefix => 'prefix/to/files')) + AS CSV + SELECT "channel", + SUM("delta") AS "changes" + FROM "wikipedia" + GROUP BY 1 + LIMIT 10 + ``` + + + + + + ```sql + INSERT INTO + EXTERN + google(bucket => 'your_bucket', prefix => 'prefix/to/files') + AS CSV + SELECT "channel", + SUM("delta") AS "changes" + FROM "wikipedia" + GROUP BY 1 + LIMIT 10 + ``` + + + + + +1. When querying, use the `rowsPerPage` query context parameter to restrict the output file size. While it's possible to add a very large LIMIT at the end of your query to force Druid to create a single file, we don't recommend this technique. + +## Learn more + +See the following topics for more information: + +* [Export to a destination](../multi-stage-query/reference.md#extern-to-export-to-a-destination) for a reference of the EXTERN. +* [SQL-based ingestion security](../multi-stage-query/security.md/#permissions-for-durable-storage) for cloud permission requirements for MSQ. diff --git a/extensions-contrib/aliyun-oss-extensions/pom.xml b/extensions-contrib/aliyun-oss-extensions/pom.xml index 82ee50eddf97..77c76c28806a 100644 --- a/extensions-contrib/aliyun-oss-extensions/pom.xml +++ b/extensions-contrib/aliyun-oss-extensions/pom.xml @@ -80,11 +80,6 @@ jackson-core provided - - com.google.inject.extensions - guice-multibindings - provided - com.google.guava guava diff --git a/extensions-contrib/aliyun-oss-extensions/src/main/java/org/apache/druid/storage/aliyun/OssDataSegmentMover.java b/extensions-contrib/aliyun-oss-extensions/src/main/java/org/apache/druid/storage/aliyun/OssDataSegmentMover.java index eff024d3e0e9..0bfd3efaff57 100644 --- a/extensions-contrib/aliyun-oss-extensions/src/main/java/org/apache/druid/storage/aliyun/OssDataSegmentMover.java +++ b/extensions-contrib/aliyun-oss-extensions/src/main/java/org/apache/druid/storage/aliyun/OssDataSegmentMover.java @@ -101,7 +101,7 @@ public DataSegment move(DataSegment segment, Map targetLoadSpec) .putAll( Maps.filterKeys( loadSpec, - new Predicate() + new Predicate<>() { @Override public boolean apply(String input) diff --git a/extensions-contrib/aliyun-oss-extensions/src/main/java/org/apache/druid/storage/aliyun/OssDataSegmentPuller.java b/extensions-contrib/aliyun-oss-extensions/src/main/java/org/apache/druid/storage/aliyun/OssDataSegmentPuller.java index ebe9b2b49439..fa8e0193b052 100644 --- a/extensions-contrib/aliyun-oss-extensions/src/main/java/org/apache/druid/storage/aliyun/OssDataSegmentPuller.java +++ b/extensions-contrib/aliyun-oss-extensions/src/main/java/org/apache/druid/storage/aliyun/OssDataSegmentPuller.java @@ -245,7 +245,7 @@ public boolean delete() public Predicate shouldRetryPredicate() { // Yay! smart retries! - return new Predicate() + return new Predicate<>() { @Override public boolean apply(Throwable e) diff --git a/extensions-contrib/aliyun-oss-extensions/src/main/java/org/apache/druid/storage/aliyun/OssUtils.java b/extensions-contrib/aliyun-oss-extensions/src/main/java/org/apache/druid/storage/aliyun/OssUtils.java index c9ec689eccaa..fa1b656f0aa9 100644 --- a/extensions-contrib/aliyun-oss-extensions/src/main/java/org/apache/druid/storage/aliyun/OssUtils.java +++ b/extensions-contrib/aliyun-oss-extensions/src/main/java/org/apache/druid/storage/aliyun/OssUtils.java @@ -57,7 +57,7 @@ static boolean isServiceExceptionRecoverable(OSSException ex) return !badStatusCode && (isIOException || isTimeout); } - public static final Predicate RETRYABLE = new Predicate() + public static final Predicate RETRYABLE = new Predicate<>() { @Override public boolean apply(Throwable e) diff --git a/extensions-contrib/cassandra-storage/pom.xml b/extensions-contrib/cassandra-storage/pom.xml index 7b5fc2be1831..5a2504bd825c 100644 --- a/extensions-contrib/cassandra-storage/pom.xml +++ b/extensions-contrib/cassandra-storage/pom.xml @@ -174,11 +174,6 @@ jackson-core provided - - com.google.inject.extensions - guice-multibindings - provided - diff --git a/extensions-contrib/cassandra-storage/src/main/java/org/apache/druid/storage/cassandra/CassandraStorage.java b/extensions-contrib/cassandra-storage/src/main/java/org/apache/druid/storage/cassandra/CassandraStorage.java index 9588eb946fb5..de03b3476a2d 100644 --- a/extensions-contrib/cassandra-storage/src/main/java/org/apache/druid/storage/cassandra/CassandraStorage.java +++ b/extensions-contrib/cassandra-storage/src/main/java/org/apache/druid/storage/cassandra/CassandraStorage.java @@ -66,7 +66,7 @@ public CassandraStorage(CassandraDataSegmentConfig config) this.config = config; indexStorage = new CassandraChunkedStorageProvider(keyspace, INDEX_TABLE_NAME); - descriptorStorage = new ColumnFamily(DESCRIPTOR_TABLE_NAME, + descriptorStorage = new ColumnFamily<>(DESCRIPTOR_TABLE_NAME, StringSerializer.get(), StringSerializer.get()); } } diff --git a/extensions-contrib/cloudfiles-extensions/pom.xml b/extensions-contrib/cloudfiles-extensions/pom.xml index 0fd5018d1311..e5076e1d2284 100644 --- a/extensions-contrib/cloudfiles-extensions/pom.xml +++ b/extensions-contrib/cloudfiles-extensions/pom.xml @@ -56,12 +56,6 @@ - - com.google.inject.extensions - guice-multibindings - provided - - commons-io commons-io diff --git a/extensions-contrib/cloudfiles-extensions/src/main/java/org/apache/druid/storage/cloudfiles/CloudFilesUtils.java b/extensions-contrib/cloudfiles-extensions/src/main/java/org/apache/druid/storage/cloudfiles/CloudFilesUtils.java index bc883b28fd84..fe903b080230 100644 --- a/extensions-contrib/cloudfiles-extensions/src/main/java/org/apache/druid/storage/cloudfiles/CloudFilesUtils.java +++ b/extensions-contrib/cloudfiles-extensions/src/main/java/org/apache/druid/storage/cloudfiles/CloudFilesUtils.java @@ -31,7 +31,7 @@ public class CloudFilesUtils { - public static final Predicate CLOUDFILESRETRY = new Predicate() + public static final Predicate CLOUDFILESRETRY = new Predicate<>() { @Override public boolean apply(Throwable e) diff --git a/extensions-contrib/compressed-bigdecimal/src/main/java/org/apache/druid/compressedbigdecimal/CompressedBigDecimalMetricSerde.java b/extensions-contrib/compressed-bigdecimal/src/main/java/org/apache/druid/compressedbigdecimal/CompressedBigDecimalMetricSerde.java index 1decd3df137a..d76896c83e09 100644 --- a/extensions-contrib/compressed-bigdecimal/src/main/java/org/apache/druid/compressedbigdecimal/CompressedBigDecimalMetricSerde.java +++ b/extensions-contrib/compressed-bigdecimal/src/main/java/org/apache/druid/compressedbigdecimal/CompressedBigDecimalMetricSerde.java @@ -49,7 +49,7 @@ public String getTypeName() @Override public ComplexMetricExtractor getExtractor() { - return new ComplexMetricExtractor() + return new ComplexMetricExtractor<>() { @Override public Class extractedClass() diff --git a/extensions-contrib/ddsketch/src/main/java/org/apache/druid/query/aggregation/ddsketch/DDSketchAggregatorFactory.java b/extensions-contrib/ddsketch/src/main/java/org/apache/druid/query/aggregation/ddsketch/DDSketchAggregatorFactory.java index 1081fdbd0ead..e70a7cc2c6cf 100644 --- a/extensions-contrib/ddsketch/src/main/java/org/apache/druid/query/aggregation/ddsketch/DDSketchAggregatorFactory.java +++ b/extensions-contrib/ddsketch/src/main/java/org/apache/druid/query/aggregation/ddsketch/DDSketchAggregatorFactory.java @@ -264,7 +264,7 @@ public AggregatorFactory withName(String newName) @Override public AggregateCombiner makeAggregateCombiner() { - return new ObjectAggregateCombiner() + return new ObjectAggregateCombiner<>() { private DDSketch combined = DDSketches.collapsingLowestDense(relativeError, numBins); diff --git a/extensions-contrib/distinctcount/src/test/java/org/apache/druid/query/aggregation/distinctcount/DistinctCountTopNQueryTest.java b/extensions-contrib/distinctcount/src/test/java/org/apache/druid/query/aggregation/distinctcount/DistinctCountTopNQueryTest.java index 7e3690cc1cb6..b54c9af1b2f7 100644 --- a/extensions-contrib/distinctcount/src/test/java/org/apache/druid/query/aggregation/distinctcount/DistinctCountTopNQueryTest.java +++ b/extensions-contrib/distinctcount/src/test/java/org/apache/druid/query/aggregation/distinctcount/DistinctCountTopNQueryTest.java @@ -60,7 +60,7 @@ public void setup() { pool = new CloseableStupidPool<>( "TopNQueryEngine-bufferPool", - new Supplier() + new Supplier<>() { @Override public ByteBuffer get() diff --git a/extensions-contrib/druid-deltalake-extensions/src/main/java/org/apache/druid/delta/input/DeltaInputSourceReader.java b/extensions-contrib/druid-deltalake-extensions/src/main/java/org/apache/druid/delta/input/DeltaInputSourceReader.java index 9ac7c253ef35..672a126f7c46 100644 --- a/extensions-contrib/druid-deltalake-extensions/src/main/java/org/apache/druid/delta/input/DeltaInputSourceReader.java +++ b/extensions-contrib/druid-deltalake-extensions/src/main/java/org/apache/druid/delta/input/DeltaInputSourceReader.java @@ -68,7 +68,7 @@ public CloseableIterator sample() { CloseableIterator inner = read(); - return new CloseableIterator() + return new CloseableIterator<>() { @Override public void close() throws IOException diff --git a/extensions-contrib/grpc-query/pom.xml b/extensions-contrib/grpc-query/pom.xml index e46f0c35fd2d..0eb714419373 100644 --- a/extensions-contrib/grpc-query/pom.xml +++ b/extensions-contrib/grpc-query/pom.xml @@ -96,11 +96,6 @@ jackson-core provided - - com.google.inject.extensions - guice-multibindings - provided - com.google.guava guava diff --git a/extensions-contrib/kafka-emitter/src/test/java/org/apache/druid/emitter/kafka/KafkaEmitterConfigTest.java b/extensions-contrib/kafka-emitter/src/test/java/org/apache/druid/emitter/kafka/KafkaEmitterConfigTest.java index 24b743a06611..0d5c5e312e84 100644 --- a/extensions-contrib/kafka-emitter/src/test/java/org/apache/druid/emitter/kafka/KafkaEmitterConfigTest.java +++ b/extensions-contrib/kafka-emitter/src/test/java/org/apache/druid/emitter/kafka/KafkaEmitterConfigTest.java @@ -91,7 +91,7 @@ public void testSerDeserKafkaEmitterConfigNullRequestTopic() throws IOException @Test public void testSerDeserKafkaEmitterConfigNullMetricsTopic() throws IOException { - Set eventTypeSet = new HashSet(); + Set eventTypeSet = new HashSet<>(); eventTypeSet.add(KafkaEmitterConfig.EventType.SEGMENT_METADATA); KafkaEmitterConfig kafkaEmitterConfig = new KafkaEmitterConfig( "hostname", diff --git a/extensions-contrib/kubernetes-overlord-extensions/pom.xml b/extensions-contrib/kubernetes-overlord-extensions/pom.xml index 55f970fb9c39..1d63a351c598 100644 --- a/extensions-contrib/kubernetes-overlord-extensions/pom.xml +++ b/extensions-contrib/kubernetes-overlord-extensions/pom.xml @@ -212,11 +212,6 @@ jackson-databind provided - - com.google.inject.extensions - guice-multibindings - provided - joda-time joda-time diff --git a/extensions-contrib/kubernetes-overlord-extensions/src/main/java/org/apache/druid/k8s/overlord/KubernetesAndWorkerTaskRunnerConfig.java b/extensions-contrib/kubernetes-overlord-extensions/src/main/java/org/apache/druid/k8s/overlord/KubernetesAndWorkerTaskRunnerConfig.java index 0ffeb0103afa..3efbfa18dce1 100644 --- a/extensions-contrib/kubernetes-overlord-extensions/src/main/java/org/apache/druid/k8s/overlord/KubernetesAndWorkerTaskRunnerConfig.java +++ b/extensions-contrib/kubernetes-overlord-extensions/src/main/java/org/apache/druid/k8s/overlord/KubernetesAndWorkerTaskRunnerConfig.java @@ -31,8 +31,8 @@ public class KubernetesAndWorkerTaskRunnerConfig { - private final String RUNNER_STRATEGY_TYPE = "runnerStrategy.type"; - private final String RUNNER_STRATEGY_WORKER_TYPE = "runnerStrategy.workerType"; + private static final String RUNNER_STRATEGY_TYPE = "runnerStrategy.type"; + private static final String RUNNER_STRATEGY_WORKER_TYPE = "runnerStrategy.workerType"; /** * Select which runner type a task would run on, options are k8s or worker. */ diff --git a/extensions-contrib/kubernetes-overlord-extensions/src/test/java/org/apache/druid/k8s/overlord/KubernetesTaskRunnerTest.java b/extensions-contrib/kubernetes-overlord-extensions/src/test/java/org/apache/druid/k8s/overlord/KubernetesTaskRunnerTest.java index a6c01ee306a5..756a4ad5a548 100644 --- a/extensions-contrib/kubernetes-overlord-extensions/src/test/java/org/apache/druid/k8s/overlord/KubernetesTaskRunnerTest.java +++ b/extensions-contrib/kubernetes-overlord-extensions/src/test/java/org/apache/druid/k8s/overlord/KubernetesTaskRunnerTest.java @@ -603,7 +603,7 @@ public TaskLocation getLocation() runner.tasks.put(task.getId(), workItem); - ListenableFuture future = new ListenableFuture() + ListenableFuture future = new ListenableFuture<>() { @Override public void addListener(Runnable runnable, Executor executor) diff --git a/extensions-contrib/materialized-view-selection/pom.xml b/extensions-contrib/materialized-view-selection/pom.xml index 4e44bfde5c6c..1cc85426d2b6 100644 --- a/extensions-contrib/materialized-view-selection/pom.xml +++ b/extensions-contrib/materialized-view-selection/pom.xml @@ -84,11 +84,6 @@ jackson-core provided - - com.google.inject.extensions - guice-multibindings - provided - org.jdbi jdbi diff --git a/extensions-contrib/materialized-view-selection/src/main/java/org/apache/druid/query/materializedview/DerivativeDataSourceManager.java b/extensions-contrib/materialized-view-selection/src/main/java/org/apache/druid/query/materializedview/DerivativeDataSourceManager.java index 3cff0da0075a..82ad698d8bda 100644 --- a/extensions-contrib/materialized-view-selection/src/main/java/org/apache/druid/query/materializedview/DerivativeDataSourceManager.java +++ b/extensions-contrib/materialized-view-selection/src/main/java/org/apache/druid/query/materializedview/DerivativeDataSourceManager.java @@ -219,7 +219,7 @@ private void updateDerivatives() private long getAvgSizePerGranularity(String datasource) { return connector.retryWithHandle( - new HandleCallback() { + new HandleCallback<>() { Set intervals = new HashSet<>(); long totalSize = 0; @Override diff --git a/extensions-contrib/materialized-view-selection/src/main/java/org/apache/druid/query/materializedview/MaterializedViewUtils.java b/extensions-contrib/materialized-view-selection/src/main/java/org/apache/druid/query/materializedview/MaterializedViewUtils.java index f28d67a62629..7c39b8fdc447 100644 --- a/extensions-contrib/materialized-view-selection/src/main/java/org/apache/druid/query/materializedview/MaterializedViewUtils.java +++ b/extensions-contrib/materialized-view-selection/src/main/java/org/apache/druid/query/materializedview/MaterializedViewUtils.java @@ -47,7 +47,7 @@ public class MaterializedViewUtils */ public static Set getRequiredFields(Query query) { - Set dimsInFilter = null == query.getFilter() ? new HashSet() : query.getFilter().getRequiredColumns(); + Set dimsInFilter = null == query.getFilter() ? new HashSet<>() : query.getFilter().getRequiredColumns(); Set dimensions = new HashSet<>(dimsInFilter); if (query instanceof TopNQuery) { diff --git a/extensions-contrib/moving-average-query/pom.xml b/extensions-contrib/moving-average-query/pom.xml index 119296634566..d948bec36867 100644 --- a/extensions-contrib/moving-average-query/pom.xml +++ b/extensions-contrib/moving-average-query/pom.xml @@ -79,11 +79,6 @@ jackson-core provided - - com.google.inject.extensions - guice-multibindings - provided - com.google.guava guava diff --git a/extensions-contrib/moving-average-query/src/main/java/org/apache/druid/query/movingaverage/MovingAverageQueryToolChest.java b/extensions-contrib/moving-average-query/src/main/java/org/apache/druid/query/movingaverage/MovingAverageQueryToolChest.java index 8d28faaa251b..19492c4a0d51 100644 --- a/extensions-contrib/moving-average-query/src/main/java/org/apache/druid/query/movingaverage/MovingAverageQueryToolChest.java +++ b/extensions-contrib/moving-average-query/src/main/java/org/apache/druid/query/movingaverage/MovingAverageQueryToolChest.java @@ -84,7 +84,7 @@ public QueryMetrics makeMetrics(MovingAverageQuery q public Function makePostComputeManipulatorFn(MovingAverageQuery query, MetricManipulationFn fn) { - return new Function() + return new Function<>() { @Override @@ -122,9 +122,7 @@ public Row apply(Row result) @Override public TypeReference getResultTypeReference() { - return new TypeReference() - { - }; + return new TypeReference<>() {}; } @Override diff --git a/extensions-contrib/moving-average-query/src/test/java/org/apache/druid/query/movingaverage/MovingAverageQueryTest.java b/extensions-contrib/moving-average-query/src/test/java/org/apache/druid/query/movingaverage/MovingAverageQueryTest.java index 68668c9b3aa3..d86b6913cfb0 100644 --- a/extensions-contrib/moving-average-query/src/test/java/org/apache/druid/query/movingaverage/MovingAverageQueryTest.java +++ b/extensions-contrib/moving-average-query/src/test/java/org/apache/druid/query/movingaverage/MovingAverageQueryTest.java @@ -234,9 +234,7 @@ private Class getExpectedQueryType() private TypeReference> getExpectedResultType() { - return new TypeReference>() - { - }; + return new TypeReference<>() {}; } /** diff --git a/extensions-contrib/moving-average-query/src/test/java/org/apache/druid/query/movingaverage/averagers/BaseAveragerFactoryTest.java b/extensions-contrib/moving-average-query/src/test/java/org/apache/druid/query/movingaverage/averagers/BaseAveragerFactoryTest.java index 30d97c81eb30..6c43e2aa7f62 100644 --- a/extensions-contrib/moving-average-query/src/test/java/org/apache/druid/query/movingaverage/averagers/BaseAveragerFactoryTest.java +++ b/extensions-contrib/moving-average-query/src/test/java/org/apache/druid/query/movingaverage/averagers/BaseAveragerFactoryTest.java @@ -33,7 +33,7 @@ public class BaseAveragerFactoryTest @Before public void setup() { - fac = new BaseAveragerFactory("test", 5, "field", 1) + fac = new BaseAveragerFactory<>("test", 5, "field", 1) { @Override public Averager createAverager() diff --git a/extensions-contrib/rabbit-stream-indexing-service/src/main/java/org/apache/druid/indexing/rabbitstream/IncrementalPublishingRabbitStreamIndexTaskRunner.java b/extensions-contrib/rabbit-stream-indexing-service/src/main/java/org/apache/druid/indexing/rabbitstream/IncrementalPublishingRabbitStreamIndexTaskRunner.java index 9265d70f992c..75663d23063f 100644 --- a/extensions-contrib/rabbit-stream-indexing-service/src/main/java/org/apache/druid/indexing/rabbitstream/IncrementalPublishingRabbitStreamIndexTaskRunner.java +++ b/extensions-contrib/rabbit-stream-indexing-service/src/main/java/org/apache/druid/indexing/rabbitstream/IncrementalPublishingRabbitStreamIndexTaskRunner.java @@ -135,8 +135,7 @@ protected boolean isEndOfShard(Long seqNum) @Override public TypeReference>> getSequenceMetadataTypeReference() { - return new TypeReference>>() { - }; + return new TypeReference<>() {}; } @Nullable @@ -149,8 +148,8 @@ protected TreeMap> getCheckPointsFromContext( log.debug("Got checkpoints from task context[%s].", checkpointsString); return toolbox.getJsonMapper().readValue( checkpointsString, - new TypeReference>>() { - }); + new TypeReference<>() {} + ); } else { return null; } diff --git a/extensions-contrib/rabbit-stream-indexing-service/src/main/java/org/apache/druid/indexing/rabbitstream/RabbitStreamRecordSupplier.java b/extensions-contrib/rabbit-stream-indexing-service/src/main/java/org/apache/druid/indexing/rabbitstream/RabbitStreamRecordSupplier.java index 7488f76dfd92..56d7cf44ea1d 100644 --- a/extensions-contrib/rabbit-stream-indexing-service/src/main/java/org/apache/druid/indexing/rabbitstream/RabbitStreamRecordSupplier.java +++ b/extensions-contrib/rabbit-stream-indexing-service/src/main/java/org/apache/druid/indexing/rabbitstream/RabbitStreamRecordSupplier.java @@ -112,7 +112,7 @@ public RabbitStreamRecordSupplier( // stateSemaphore protects isRunning and consumers stateSemaphore = new Semaphore(1, true); isRunning = false; - consumers = new ArrayList(); + consumers = new ArrayList<>(); this.password = null; this.username = null; @@ -238,7 +238,7 @@ private void removeOldAssignments(Set> streamPartitionst Iterator> streamBuilderIterator = streamBuilders.entrySet().iterator(); while (streamBuilderIterator.hasNext()) { Map.Entry entry = streamBuilderIterator.next(); - StreamPartition comparitor = new StreamPartition(getStreamFromSubstream(entry.getKey()), entry.getKey()); + StreamPartition comparitor = new StreamPartition<>(getStreamFromSubstream(entry.getKey()), entry.getKey()); if (!streamPartitionstoKeep.contains(comparitor)) { streamBuilderIterator.remove(); } @@ -247,7 +247,7 @@ private void removeOldAssignments(Set> streamPartitionst Iterator> offsetIterator = offsetMap.entrySet().iterator(); while (offsetIterator.hasNext()) { Map.Entry entry = offsetIterator.next(); - StreamPartition comparitor = new StreamPartition(getStreamFromSubstream(entry.getKey()), entry.getKey()); + StreamPartition comparitor = new StreamPartition<>(getStreamFromSubstream(entry.getKey()), entry.getKey()); if (!streamPartitionstoKeep.contains(comparitor)) { offsetIterator.remove(); } diff --git a/extensions-contrib/rabbit-stream-indexing-service/src/main/java/org/apache/druid/indexing/rabbitstream/supervisor/RabbitStreamSupervisor.java b/extensions-contrib/rabbit-stream-indexing-service/src/main/java/org/apache/druid/indexing/rabbitstream/supervisor/RabbitStreamSupervisor.java index 5e9d3c3d2c6a..9ce9bd331cfc 100644 --- a/extensions-contrib/rabbit-stream-indexing-service/src/main/java/org/apache/druid/indexing/rabbitstream/supervisor/RabbitStreamSupervisor.java +++ b/extensions-contrib/rabbit-stream-indexing-service/src/main/java/org/apache/druid/indexing/rabbitstream/supervisor/RabbitStreamSupervisor.java @@ -83,8 +83,7 @@ */ public class RabbitStreamSupervisor extends SeekableStreamSupervisor { - public static final TypeReference>> CHECKPOINTS_TYPE_REF = new TypeReference>>() { - }; + public static final TypeReference>> CHECKPOINTS_TYPE_REF = new TypeReference<>() {}; private static final EmittingLogger log = new EmittingLogger(RabbitStreamSupervisor.class); private static final Long NOT_SET = -1L; diff --git a/extensions-contrib/spectator-histogram/src/test/java/org/apache/druid/spectator/histogram/SpectatorHistogramTest.java b/extensions-contrib/spectator-histogram/src/test/java/org/apache/druid/spectator/histogram/SpectatorHistogramTest.java index 77b81f398477..ac0d59189dbd 100644 --- a/extensions-contrib/spectator-histogram/src/test/java/org/apache/druid/spectator/histogram/SpectatorHistogramTest.java +++ b/extensions-contrib/spectator-histogram/src/test/java/org/apache/druid/spectator/histogram/SpectatorHistogramTest.java @@ -281,7 +281,7 @@ public void testSerializeAndDeserialize() throws IOException histogram.add(32, 4294967295L); histogram.add(33, 4294967296L); - ColumnValueSelector selector = new ColumnValueSelector() + ColumnValueSelector selector = new ColumnValueSelector<>() { private int callCount = 0; diff --git a/extensions-contrib/sqlserver-metadata-storage/pom.xml b/extensions-contrib/sqlserver-metadata-storage/pom.xml index 1fd186f9dc84..37e39c3f5eae 100644 --- a/extensions-contrib/sqlserver-metadata-storage/pom.xml +++ b/extensions-contrib/sqlserver-metadata-storage/pom.xml @@ -67,11 +67,6 @@ jackson-databind provided - - com.google.inject.extensions - guice-multibindings - provided - org.apache.commons commons-dbcp2 diff --git a/extensions-contrib/sqlserver-metadata-storage/src/main/java/org/apache/druid/metadata/storage/sqlserver/SQLServerConnector.java b/extensions-contrib/sqlserver-metadata-storage/src/main/java/org/apache/druid/metadata/storage/sqlserver/SQLServerConnector.java index 42787a0733cd..7b3c94f49926 100644 --- a/extensions-contrib/sqlserver-metadata-storage/src/main/java/org/apache/druid/metadata/storage/sqlserver/SQLServerConnector.java +++ b/extensions-contrib/sqlserver-metadata-storage/src/main/java/org/apache/druid/metadata/storage/sqlserver/SQLServerConnector.java @@ -240,7 +240,7 @@ public Void insertOrUpdate( final byte[] value) { return getDBI().withHandle( - new HandleCallback() + new HandleCallback<>() { @Override public Void withHandle(Handle handle) diff --git a/extensions-contrib/statsd-emitter/src/main/resources/defaultMetricDimensions.json b/extensions-contrib/statsd-emitter/src/main/resources/defaultMetricDimensions.json index a83aeb1bc7e1..48bc72505fe5 100644 --- a/extensions-contrib/statsd-emitter/src/main/resources/defaultMetricDimensions.json +++ b/extensions-contrib/statsd-emitter/src/main/resources/defaultMetricDimensions.json @@ -190,7 +190,7 @@ "namespace/cache/numEntries" : { "dimensions" : [], "type" : "gauge" }, "namespace/cache/heapSizeInBytes" : { "dimensions" : [], "type" : "gauge" }, - "service/heartbeat" : { "dimensions" : ["leader"], "type" : "count" }, + "service/heartbeat" : { "dimensions" : ["leader", "workerVersion", "category", "status", "taskId", "groupId", "dataSource", "taskStatus" ], "type" : "count" }, "killTask/availableSlot/count" : { "dimensions" : [], "type" : "count" }, "killTask/maxSlot/count" : { "dimensions" : [], "type" : "count" }, diff --git a/extensions-contrib/thrift-extensions/src/main/java/org/apache/druid/data/input/thrift/ThriftDeserialization.java b/extensions-contrib/thrift-extensions/src/main/java/org/apache/druid/data/input/thrift/ThriftDeserialization.java index 90b802788499..956d9f3c8ce7 100644 --- a/extensions-contrib/thrift-extensions/src/main/java/org/apache/druid/data/input/thrift/ThriftDeserialization.java +++ b/extensions-contrib/thrift-extensions/src/main/java/org/apache/druid/data/input/thrift/ThriftDeserialization.java @@ -39,7 +39,7 @@ public class ThriftDeserialization private static final Logger log = LoggerFactory.getLogger(ThriftDeserialization.class); - private static final ThreadLocal DESERIALIZER_COMPACT = new ThreadLocal() + private static final ThreadLocal DESERIALIZER_COMPACT = new ThreadLocal<>() { @Override protected TDeserializer initialValue() @@ -48,7 +48,7 @@ protected TDeserializer initialValue() } }; - private static final ThreadLocal DESERIALIZER_BINARY = new ThreadLocal() + private static final ThreadLocal DESERIALIZER_BINARY = new ThreadLocal<>() { @Override protected TDeserializer initialValue() @@ -57,7 +57,7 @@ protected TDeserializer initialValue() } }; - private static final ThreadLocal DESERIALIZER_JSON = new ThreadLocal() + private static final ThreadLocal DESERIALIZER_JSON = new ThreadLocal<>() { @Override protected TDeserializer initialValue() @@ -66,7 +66,7 @@ protected TDeserializer initialValue() } }; - public static final ThreadLocal SERIALIZER_SIMPLE_JSON = new ThreadLocal() + public static final ThreadLocal SERIALIZER_SIMPLE_JSON = new ThreadLocal<>() { @Override protected TSerializer initialValue() diff --git a/extensions-contrib/virtual-columns/src/test/java/org/apache/druid/segment/MapVirtualColumnTestBase.java b/extensions-contrib/virtual-columns/src/test/java/org/apache/druid/segment/MapVirtualColumnTestBase.java index 29270c0df734..c236ecb45ae0 100644 --- a/extensions-contrib/virtual-columns/src/test/java/org/apache/druid/segment/MapVirtualColumnTestBase.java +++ b/extensions-contrib/virtual-columns/src/test/java/org/apache/druid/segment/MapVirtualColumnTestBase.java @@ -62,7 +62,7 @@ static IncrementalIndex generateIndex() throws IOException .withMinTimestamp(DateTimes.of("2011-01-12T00:00:00.000Z").getMillis()) .build(); - return TestIndex.loadIncrementalIndex( + return TestIndex.loadIncrementalIndexFromCharSource( () -> new OnheapIncrementalIndex.Builder() .setIndexSchema(schema) .setMaxRowCount(10000) diff --git a/extensions-core/avro-extensions/src/main/java/org/apache/druid/data/input/avro/AvroOCFReader.java b/extensions-core/avro-extensions/src/main/java/org/apache/druid/data/input/avro/AvroOCFReader.java index 012e2e801a0d..a071f38abb76 100644 --- a/extensions-core/avro-extensions/src/main/java/org/apache/druid/data/input/avro/AvroOCFReader.java +++ b/extensions-core/avro-extensions/src/main/java/org/apache/druid/data/input/avro/AvroOCFReader.java @@ -92,7 +92,7 @@ protected CloseableIterator intermediateRowIterator() throws IOEx datumReader.setExpected(readerSchema); closer.register(dataFileReader); - return new CloseableIterator() + return new CloseableIterator<>() { @Override public boolean hasNext() diff --git a/extensions-core/avro-extensions/src/test/java/org/apache/druid/data/input/AvroStreamInputRowParserTest.java b/extensions-core/avro-extensions/src/test/java/org/apache/druid/data/input/AvroStreamInputRowParserTest.java index f0cfd372cb84..a7ad60b13064 100644 --- a/extensions-core/avro-extensions/src/test/java/org/apache/druid/data/input/AvroStreamInputRowParserTest.java +++ b/extensions-core/avro-extensions/src/test/java/org/apache/druid/data/input/AvroStreamInputRowParserTest.java @@ -136,7 +136,7 @@ public class AvroStreamInputRowParserTest private static final List SOME_STRING_ARRAY_VALUE = Arrays.asList("8", "4", "2", "1", null); private static final List SOME_INT_ARRAY_VALUE = Arrays.asList(1, 2, 4, 8); static final Map SOME_INT_VALUE_MAP_VALUE = Maps.asMap( - new HashSet<>(Arrays.asList("8", "2", "4", "1")), new Function() + new HashSet<>(Arrays.asList("8", "2", "4", "1")), new Function<>() { @Nonnull @Override @@ -147,7 +147,7 @@ public Integer apply(@Nullable CharSequence input) } ); static final Map SOME_STRING_VALUE_MAP_VALUE = Maps.asMap( - new HashSet<>(Arrays.asList("8", "2", "4", "1")), new Function() + new HashSet<>(Arrays.asList("8", "2", "4", "1")), new Function<>() { @Nonnull @Override @@ -429,7 +429,7 @@ static void assertInputRowCorrect(InputRow inputRow, List expectedDimens .on(",") .withKeyValueSeparator("=") .split(BRACES_AND_SPACE.matcher(inputRow.getDimension("someIntValueMap").get(0)).replaceAll("")), - new Function() + new Function<>() { @Nullable @Override diff --git a/extensions-core/avro-extensions/src/test/java/org/apache/druid/data/input/avro/AvroOCFInputFormatTest.java b/extensions-core/avro-extensions/src/test/java/org/apache/druid/data/input/avro/AvroOCFInputFormatTest.java index ceb394b0b3b3..c5caaa387ca2 100644 --- a/extensions-core/avro-extensions/src/test/java/org/apache/druid/data/input/avro/AvroOCFInputFormatTest.java +++ b/extensions-core/avro-extensions/src/test/java/org/apache/druid/data/input/avro/AvroOCFInputFormatTest.java @@ -87,9 +87,7 @@ public void testSerdeNonDefaults() throws Exception + " ]\n" + "}"; - TypeReference> typeRef = new TypeReference>() - { - }; + TypeReference> typeRef = new TypeReference<>() {}; final Map readerSchema = jsonMapper.readValue(schemaStr, typeRef); AvroOCFInputFormat inputFormat = new AvroOCFInputFormat( jsonMapper, diff --git a/extensions-core/avro-extensions/src/test/java/org/apache/druid/data/input/avro/AvroOCFReaderTest.java b/extensions-core/avro-extensions/src/test/java/org/apache/druid/data/input/avro/AvroOCFReaderTest.java index b00c1e4ad7a1..9ee52c2ce43d 100644 --- a/extensions-core/avro-extensions/src/test/java/org/apache/druid/data/input/avro/AvroOCFReaderTest.java +++ b/extensions-core/avro-extensions/src/test/java/org/apache/druid/data/input/avro/AvroOCFReaderTest.java @@ -84,9 +84,7 @@ public void testParseWithReaderSchema() throws Exception + " ]\n" + "}"; - TypeReference> typeRef = new TypeReference>() - { - }; + TypeReference> typeRef = new TypeReference<>() {}; final Map readerSchema = mapper.readValue(schemaStr, typeRef); final InputEntityReader reader = createReader(mapper, readerSchema); @@ -114,9 +112,7 @@ public void testParseWithReaderSchemaAlias() throws Exception + " ]\n" + "}"; - TypeReference> typeRef = new TypeReference>() - { - }; + TypeReference> typeRef = new TypeReference<>() {}; final Map readerSchema = mapper.readValue(schemaStr, typeRef); final InputEntityReader reader = createReader(mapper, readerSchema); diff --git a/extensions-core/azure-extensions/pom.xml b/extensions-core/azure-extensions/pom.xml index af36456bbe70..990913c0328d 100644 --- a/extensions-core/azure-extensions/pom.xml +++ b/extensions-core/azure-extensions/pom.xml @@ -111,11 +111,6 @@ jackson-core provided - - com.google.inject.extensions - guice-multibindings - provided - com.google.guava guava diff --git a/extensions-core/azure-extensions/src/main/java/org/apache/druid/storage/azure/output/AzureStorageConnector.java b/extensions-core/azure-extensions/src/main/java/org/apache/druid/storage/azure/output/AzureStorageConnector.java index 2d6276c5a6d8..bc762ebea2cc 100644 --- a/extensions-core/azure-extensions/src/main/java/org/apache/druid/storage/azure/output/AzureStorageConnector.java +++ b/extensions-core/azure-extensions/src/main/java/org/apache/druid/storage/azure/output/AzureStorageConnector.java @@ -86,7 +86,7 @@ public ChunkingStorageConnectorParameters buildInputParams(Stri objectPath(path) )); parameters.objectOpenFunction( - new ObjectOpenFunction() + new ObjectOpenFunction<>() { @Override public InputStream open(AzureInputRange inputRange) throws IOException diff --git a/extensions-core/datasketches/src/main/java/org/apache/druid/query/aggregation/datasketches/kll/KllDoublesSketchAggregatorFactory.java b/extensions-core/datasketches/src/main/java/org/apache/druid/query/aggregation/datasketches/kll/KllDoublesSketchAggregatorFactory.java index 267953e23647..8ceedd6c5980 100644 --- a/extensions-core/datasketches/src/main/java/org/apache/druid/query/aggregation/datasketches/kll/KllDoublesSketchAggregatorFactory.java +++ b/extensions-core/datasketches/src/main/java/org/apache/druid/query/aggregation/datasketches/kll/KllDoublesSketchAggregatorFactory.java @@ -158,7 +158,7 @@ public VectorAggregator factorizeVector(VectorColumnSelectorFactory selectorFact { return ColumnProcessors.makeVectorProcessor( getFieldName(), - new VectorColumnProcessorFactory() + new VectorColumnProcessorFactory<>() { @Override public VectorAggregator makeSingleValueDimensionProcessor( diff --git a/extensions-core/datasketches/src/main/java/org/apache/druid/query/aggregation/datasketches/kll/KllDoublesSketchMergeAggregatorFactory.java b/extensions-core/datasketches/src/main/java/org/apache/druid/query/aggregation/datasketches/kll/KllDoublesSketchMergeAggregatorFactory.java index dbcd34b26c6c..5598a9646248 100644 --- a/extensions-core/datasketches/src/main/java/org/apache/druid/query/aggregation/datasketches/kll/KllDoublesSketchMergeAggregatorFactory.java +++ b/extensions-core/datasketches/src/main/java/org/apache/druid/query/aggregation/datasketches/kll/KllDoublesSketchMergeAggregatorFactory.java @@ -59,7 +59,7 @@ public Aggregator factorize(final ColumnSelectorFactory metricFactory) { final ColumnValueSelector selector = metricFactory.makeColumnValueSelector(getFieldName()); if (selector instanceof NilColumnValueSelector) { - return new KllSketchNoOpAggregator(KllDoublesSketchOperations.EMPTY_SKETCH); + return new KllSketchNoOpAggregator<>(KllDoublesSketchOperations.EMPTY_SKETCH); } return getMergeAggregator(selector); } @@ -69,7 +69,7 @@ public BufferAggregator factorizeBuffered(final ColumnSelectorFactory metricFact { final ColumnValueSelector selector = metricFactory.makeColumnValueSelector(getFieldName()); if (selector instanceof NilColumnValueSelector) { - return new KllSketchNoOpBufferAggregator(KllDoublesSketchOperations.EMPTY_SKETCH); + return new KllSketchNoOpBufferAggregator<>(KllDoublesSketchOperations.EMPTY_SKETCH); } return getMergeBufferAggregator(selector); } diff --git a/extensions-core/datasketches/src/main/java/org/apache/druid/query/aggregation/datasketches/kll/KllFloatsSketchAggregatorFactory.java b/extensions-core/datasketches/src/main/java/org/apache/druid/query/aggregation/datasketches/kll/KllFloatsSketchAggregatorFactory.java index bdd672ab1257..e60d2541d568 100644 --- a/extensions-core/datasketches/src/main/java/org/apache/druid/query/aggregation/datasketches/kll/KllFloatsSketchAggregatorFactory.java +++ b/extensions-core/datasketches/src/main/java/org/apache/druid/query/aggregation/datasketches/kll/KllFloatsSketchAggregatorFactory.java @@ -158,7 +158,7 @@ public VectorAggregator factorizeVector(VectorColumnSelectorFactory selectorFact { return ColumnProcessors.makeVectorProcessor( getFieldName(), - new VectorColumnProcessorFactory() + new VectorColumnProcessorFactory<>() { @Override public VectorAggregator makeSingleValueDimensionProcessor( diff --git a/extensions-core/datasketches/src/main/java/org/apache/druid/query/aggregation/datasketches/kll/KllFloatsSketchMergeAggregatorFactory.java b/extensions-core/datasketches/src/main/java/org/apache/druid/query/aggregation/datasketches/kll/KllFloatsSketchMergeAggregatorFactory.java index e1b0c2d6244c..c1b470ea1739 100644 --- a/extensions-core/datasketches/src/main/java/org/apache/druid/query/aggregation/datasketches/kll/KllFloatsSketchMergeAggregatorFactory.java +++ b/extensions-core/datasketches/src/main/java/org/apache/druid/query/aggregation/datasketches/kll/KllFloatsSketchMergeAggregatorFactory.java @@ -59,7 +59,7 @@ public Aggregator factorize(final ColumnSelectorFactory metricFactory) { final ColumnValueSelector selector = metricFactory.makeColumnValueSelector(getFieldName()); if (selector instanceof NilColumnValueSelector) { - return new KllSketchNoOpAggregator(KllFloatsSketchOperations.EMPTY_SKETCH); + return new KllSketchNoOpAggregator<>(KllFloatsSketchOperations.EMPTY_SKETCH); } return getMergeAggregator(selector); } @@ -69,7 +69,7 @@ public BufferAggregator factorizeBuffered(final ColumnSelectorFactory metricFact { final ColumnValueSelector selector = metricFactory.makeColumnValueSelector(getFieldName()); if (selector instanceof NilColumnValueSelector) { - return new KllSketchNoOpBufferAggregator(KllFloatsSketchOperations.EMPTY_SKETCH); + return new KllSketchNoOpBufferAggregator<>(KllFloatsSketchOperations.EMPTY_SKETCH); } return getMergeBufferAggregator(selector); } diff --git a/extensions-core/datasketches/src/main/java/org/apache/druid/query/aggregation/datasketches/kll/KllSketchAggregatorFactory.java b/extensions-core/datasketches/src/main/java/org/apache/druid/query/aggregation/datasketches/kll/KllSketchAggregatorFactory.java index 04b6d2d23031..98b46511f062 100644 --- a/extensions-core/datasketches/src/main/java/org/apache/druid/query/aggregation/datasketches/kll/KllSketchAggregatorFactory.java +++ b/extensions-core/datasketches/src/main/java/org/apache/druid/query/aggregation/datasketches/kll/KllSketchAggregatorFactory.java @@ -80,13 +80,13 @@ public Aggregator factorize(final ColumnSelectorFactory metricFactory) && metricFactory.getColumnCapabilities(fieldName).isNumeric()) { final ColumnValueSelector selector = metricFactory.makeColumnValueSelector(fieldName); if (selector instanceof NilColumnValueSelector) { - return new KllSketchNoOpAggregator(getEmptySketch()); + return new KllSketchNoOpAggregator<>(getEmptySketch()); } return getBuildAggregator(selector); } final ColumnValueSelector selector = metricFactory.makeColumnValueSelector(fieldName); if (selector instanceof NilColumnValueSelector) { - return new KllSketchNoOpAggregator(getEmptySketch()); + return new KllSketchNoOpAggregator<>(getEmptySketch()); } return getMergeAggregator(selector); } @@ -98,13 +98,13 @@ public BufferAggregator factorizeBuffered(final ColumnSelectorFactory metricFact && metricFactory.getColumnCapabilities(fieldName).isNumeric()) { final ColumnValueSelector selector = metricFactory.makeColumnValueSelector(fieldName); if (selector instanceof NilColumnValueSelector) { - return new KllSketchNoOpBufferAggregator(getEmptySketch()); + return new KllSketchNoOpBufferAggregator<>(getEmptySketch()); } return getBuildBufferAggregator(selector); } final ColumnValueSelector selector = metricFactory.makeColumnValueSelector(fieldName); if (selector instanceof NilColumnValueSelector) { - return new KllSketchNoOpBufferAggregator(getEmptySketch()); + return new KllSketchNoOpBufferAggregator<>(getEmptySketch()); } return getMergeBufferAggregator(selector); } diff --git a/extensions-core/datasketches/src/main/java/org/apache/druid/query/aggregation/datasketches/kll/KllSketchMergeAggregatorFactory.java b/extensions-core/datasketches/src/main/java/org/apache/druid/query/aggregation/datasketches/kll/KllSketchMergeAggregatorFactory.java index 86b9b507e0ae..9e9d2a98d34a 100644 --- a/extensions-core/datasketches/src/main/java/org/apache/druid/query/aggregation/datasketches/kll/KllSketchMergeAggregatorFactory.java +++ b/extensions-core/datasketches/src/main/java/org/apache/druid/query/aggregation/datasketches/kll/KllSketchMergeAggregatorFactory.java @@ -53,7 +53,7 @@ public Aggregator factorize(final ColumnSelectorFactory metricFactory) { final ColumnValueSelector selector = metricFactory.makeColumnValueSelector(getFieldName()); if (selector instanceof NilColumnValueSelector) { - return new KllSketchNoOpAggregator(getEmptySketch()); + return new KllSketchNoOpAggregator<>(getEmptySketch()); } return getMergeAggregator(selector); } @@ -63,7 +63,7 @@ public BufferAggregator factorizeBuffered(final ColumnSelectorFactory metricFact { final ColumnValueSelector selector = metricFactory.makeColumnValueSelector(getFieldName()); if (selector instanceof NilColumnValueSelector) { - return new KllSketchNoOpBufferAggregator(getEmptySketch()); + return new KllSketchNoOpBufferAggregator<>(getEmptySketch()); } return getMergeBufferAggregator(selector); } diff --git a/extensions-core/datasketches/src/main/java/org/apache/druid/query/aggregation/datasketches/quantiles/DoublesSketchAggregatorFactory.java b/extensions-core/datasketches/src/main/java/org/apache/druid/query/aggregation/datasketches/quantiles/DoublesSketchAggregatorFactory.java index acdef51178fd..2d8751515426 100644 --- a/extensions-core/datasketches/src/main/java/org/apache/druid/query/aggregation/datasketches/quantiles/DoublesSketchAggregatorFactory.java +++ b/extensions-core/datasketches/src/main/java/org/apache/druid/query/aggregation/datasketches/quantiles/DoublesSketchAggregatorFactory.java @@ -170,7 +170,7 @@ public VectorAggregator factorizeVector(VectorColumnSelectorFactory selectorFact { return ColumnProcessors.makeVectorProcessor( fieldName, - new VectorColumnProcessorFactory() + new VectorColumnProcessorFactory<>() { @Override public VectorAggregator makeSingleValueDimensionProcessor( diff --git a/extensions-core/datasketches/src/main/java/org/apache/druid/query/aggregation/datasketches/theta/SketchHolder.java b/extensions-core/datasketches/src/main/java/org/apache/druid/query/aggregation/datasketches/theta/SketchHolder.java index 0d1ac2f1becf..c9229042763a 100644 --- a/extensions-core/datasketches/src/main/java/org/apache/druid/query/aggregation/datasketches/theta/SketchHolder.java +++ b/extensions-core/datasketches/src/main/java/org/apache/druid/query/aggregation/datasketches/theta/SketchHolder.java @@ -52,7 +52,7 @@ public class SketchHolder ); public static final Comparator COMPARATOR = Ordering.from( - new Comparator() + new Comparator<>() { @Override public int compare(Object o1, Object o2) @@ -81,7 +81,7 @@ public int compare(Object o1, Object o2) } ).nullsFirst(); - private static final Comparator SKETCH_COMPARATOR = new Comparator() + private static final Comparator SKETCH_COMPARATOR = new Comparator<>() { @Override public int compare(Sketch o1, Sketch o2) @@ -90,7 +90,7 @@ public int compare(Sketch o1, Sketch o2) } }; - private static final Comparator MEMORY_COMPARATOR = new Comparator() + private static final Comparator MEMORY_COMPARATOR = new Comparator<>() { @SuppressWarnings("SubtractionInCompareTo") @Override diff --git a/extensions-core/datasketches/src/main/java/org/apache/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchBuildAggregator.java b/extensions-core/datasketches/src/main/java/org/apache/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchBuildAggregator.java index 11ad6b218e1e..d0cde7c85f5c 100644 --- a/extensions-core/datasketches/src/main/java/org/apache/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchBuildAggregator.java +++ b/extensions-core/datasketches/src/main/java/org/apache/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchBuildAggregator.java @@ -50,7 +50,7 @@ public class ArrayOfDoublesSketchBuildAggregator implements Aggregator private final int nominalEntries; private final boolean canLookupUtf8; private final boolean canCacheById; - private final LinkedHashMap stringCache = new LinkedHashMap() + private final LinkedHashMap stringCache = new LinkedHashMap<>() { @Override protected boolean removeEldestEntry(Map.Entry eldest) diff --git a/extensions-core/datasketches/src/main/java/org/apache/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchBuildBufferAggregator.java b/extensions-core/datasketches/src/main/java/org/apache/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchBuildBufferAggregator.java index b925220c89fd..8acaef908191 100644 --- a/extensions-core/datasketches/src/main/java/org/apache/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchBuildBufferAggregator.java +++ b/extensions-core/datasketches/src/main/java/org/apache/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchBuildBufferAggregator.java @@ -53,7 +53,7 @@ public class ArrayOfDoublesSketchBuildBufferAggregator implements BufferAggregat private final boolean canLookupUtf8; private final boolean canCacheById; - private final LinkedHashMap stringCache = new LinkedHashMap() + private final LinkedHashMap stringCache = new LinkedHashMap<>() { @Override protected boolean removeEldestEntry(Map.Entry eldest) diff --git a/extensions-core/datasketches/src/test/java/org/apache/druid/query/aggregation/datasketches/quantiles/sql/DoublesSketchSqlAggregatorTest.java b/extensions-core/datasketches/src/test/java/org/apache/druid/query/aggregation/datasketches/quantiles/sql/DoublesSketchSqlAggregatorTest.java index ba1eb347fe71..9cc90617347d 100644 --- a/extensions-core/datasketches/src/test/java/org/apache/druid/query/aggregation/datasketches/quantiles/sql/DoublesSketchSqlAggregatorTest.java +++ b/extensions-core/datasketches/src/test/java/org/apache/druid/query/aggregation/datasketches/quantiles/sql/DoublesSketchSqlAggregatorTest.java @@ -27,6 +27,7 @@ import org.apache.druid.java.util.common.Intervals; import org.apache.druid.java.util.common.granularity.Granularities; import org.apache.druid.query.Druids; +import org.apache.druid.query.JoinAlgorithm; import org.apache.druid.query.JoinDataSource; import org.apache.druid.query.QueryContexts; import org.apache.druid.query.QueryDataSource; @@ -390,7 +391,8 @@ public void testSubqueryWithNestedGroupBy() JoinType.INNER, null, TestExprMacroTable.INSTANCE, - null + null, + JoinAlgorithm.BROADCAST ) ) .intervals(querySegmentSpec(Intervals.ETERNITY)) diff --git a/extensions-core/datasketches/src/test/java/org/apache/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchAggregatorFactoryTest.java b/extensions-core/datasketches/src/test/java/org/apache/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchAggregatorFactoryTest.java index d62c6c61ce7a..1009391c9574 100644 --- a/extensions-core/datasketches/src/test/java/org/apache/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchAggregatorFactoryTest.java +++ b/extensions-core/datasketches/src/test/java/org/apache/druid/query/aggregation/datasketches/tuple/ArrayOfDoublesSketchAggregatorFactoryTest.java @@ -56,7 +56,9 @@ public void makeAggregateCombiner() sketch2.update("b", new double[] {1}); sketch2.update("c", new double[] {1}); - TestObjectColumnSelector selector = new TestObjectColumnSelector(new ArrayOfDoublesSketch[] {sketch1, sketch2}); + TestObjectColumnSelector selector = new TestObjectColumnSelector<>( + new ArrayOfDoublesSketch[]{sketch1, sketch2} + ); combiner.reset(selector); Assert.assertEquals(1, combiner.getObject().getEstimate(), 0); diff --git a/extensions-core/druid-basic-security/src/main/java/org/apache/druid/security/basic/BasicAuthUtils.java b/extensions-core/druid-basic-security/src/main/java/org/apache/druid/security/basic/BasicAuthUtils.java index 5169ae56d8e6..44948df4a6ad 100644 --- a/extensions-core/druid-basic-security/src/main/java/org/apache/druid/security/basic/BasicAuthUtils.java +++ b/extensions-core/druid-basic-security/src/main/java/org/apache/druid/security/basic/BasicAuthUtils.java @@ -63,34 +63,22 @@ public class BasicAuthUtils (throwable) -> throwable instanceof BasicSecurityDBResourceException; public static final TypeReference> AUTHENTICATOR_USER_MAP_TYPE_REFERENCE = - new TypeReference>() - { - }; + new TypeReference<>() {}; public static final TypeReference> AUTHORIZER_USER_MAP_TYPE_REFERENCE = - new TypeReference>() - { - }; + new TypeReference<>() {}; public static final TypeReference> AUTHORIZER_GROUP_MAPPING_MAP_TYPE_REFERENCE = - new TypeReference>() - { - }; + new TypeReference<>() {}; public static final TypeReference> AUTHORIZER_ROLE_MAP_TYPE_REFERENCE = - new TypeReference>() - { - }; + new TypeReference<>() {}; public static final TypeReference AUTHORIZER_USER_AND_ROLE_MAP_TYPE_REFERENCE = - new TypeReference() - { - }; + new TypeReference<>() {}; public static final TypeReference AUTHORIZER_GROUP_MAPPING_AND_ROLE_MAP_TYPE_REFERENCE = - new TypeReference() - { - }; + new TypeReference<>() {}; public static byte[] generateSalt() { diff --git a/extensions-core/druid-basic-security/src/main/java/org/apache/druid/security/basic/authentication/db/updater/CoordinatorBasicAuthenticatorMetadataStorageUpdater.java b/extensions-core/druid-basic-security/src/main/java/org/apache/druid/security/basic/authentication/db/updater/CoordinatorBasicAuthenticatorMetadataStorageUpdater.java index a83c20e422d4..bbebdf9e7282 100644 --- a/extensions-core/druid-basic-security/src/main/java/org/apache/druid/security/basic/authentication/db/updater/CoordinatorBasicAuthenticatorMetadataStorageUpdater.java +++ b/extensions-core/druid-basic-security/src/main/java/org/apache/druid/security/basic/authentication/db/updater/CoordinatorBasicAuthenticatorMetadataStorageUpdater.java @@ -74,7 +74,7 @@ public class CoordinatorBasicAuthenticatorMetadataStorageUpdater implements Basi private final BasicAuthCommonCacheConfig commonCacheConfig; private final ObjectMapper objectMapper; private final BasicAuthenticatorCacheNotifier cacheNotifier; - private final int numRetries = 5; + private static final int NUM_RETRIES = 5; private final Map cachedUserMaps; private final Set authenticatorPrefixes; @@ -168,7 +168,7 @@ public void start() exec, new Duration(commonCacheConfig.getPollingPeriod()), new Duration(commonCacheConfig.getPollingPeriod()), - new Callable() + new Callable<>() { @Override public ScheduledExecutors.Signal call() @@ -294,7 +294,7 @@ private static String getPrefixedKeyColumn(String keyPrefix, String keyName) private void createUserInternal(String prefix, String userName) { int attempts = 0; - while (attempts < numRetries) { + while (attempts < NUM_RETRIES) { if (createUserOnce(prefix, userName)) { return; } else { @@ -308,7 +308,7 @@ private void createUserInternal(String prefix, String userName) private void deleteUserInternal(String prefix, String userName) { int attempts = 0; - while (attempts < numRetries) { + while (attempts < NUM_RETRIES) { if (deleteUserOnce(prefix, userName)) { return; } else { @@ -349,7 +349,7 @@ private void setUserCredentialsInternal(String prefix, String userName, BasicAut } int attempts = 0; - while (attempts < numRetries) { + while (attempts < NUM_RETRIES) { if (setUserCredentialOnce(prefix, userName, credentials)) { return; } else { diff --git a/extensions-core/druid-basic-security/src/main/java/org/apache/druid/security/basic/authorization/db/updater/CoordinatorBasicAuthorizerMetadataStorageUpdater.java b/extensions-core/druid-basic-security/src/main/java/org/apache/druid/security/basic/authorization/db/updater/CoordinatorBasicAuthorizerMetadataStorageUpdater.java index 2de3ef11cc87..baff948331af 100644 --- a/extensions-core/druid-basic-security/src/main/java/org/apache/druid/security/basic/authorization/db/updater/CoordinatorBasicAuthorizerMetadataStorageUpdater.java +++ b/extensions-core/druid-basic-security/src/main/java/org/apache/druid/security/basic/authorization/db/updater/CoordinatorBasicAuthorizerMetadataStorageUpdater.java @@ -91,7 +91,7 @@ public class CoordinatorBasicAuthorizerMetadataStorageUpdater implements BasicAu private final BasicAuthorizerCacheNotifier cacheNotifier; private final BasicAuthCommonCacheConfig commonCacheConfig; private final ObjectMapper objectMapper; - private final int numRetries = 5; + private static final int numRetries = 5; private final Map cachedUserMaps; private final Map cachedGroupMappingMaps; diff --git a/extensions-core/druid-basic-security/src/test/java/org/apache/druid/security/authentication/validator/LDAPCredentialsValidatorTest.java b/extensions-core/druid-basic-security/src/test/java/org/apache/druid/security/authentication/validator/LDAPCredentialsValidatorTest.java index 7a1ae2cc917b..19e35b0f8e6e 100644 --- a/extensions-core/druid-basic-security/src/test/java/org/apache/druid/security/authentication/validator/LDAPCredentialsValidatorTest.java +++ b/extensions-core/druid-basic-security/src/test/java/org/apache/druid/security/authentication/validator/LDAPCredentialsValidatorTest.java @@ -114,7 +114,7 @@ public Context getInitialContext(Hashtable environment) throws NamingExcep ArgumentMatchers.eq(LDAP_CONFIG.getBaseDn()), ArgumentMatchers.eq(StringUtils.format(LDAP_CONFIG.getUserSearch(), encodedUsername)), ArgumentMatchers.any(SearchControls.class)) - ).thenReturn(new NamingEnumeration() + ).thenReturn(new NamingEnumeration<>() { @Override public SearchResult next() diff --git a/extensions-core/druid-bloom-filter/src/test/java/org/apache/druid/query/aggregation/bloom/BloomFilterAggregatorTest.java b/extensions-core/druid-bloom-filter/src/test/java/org/apache/druid/query/aggregation/bloom/BloomFilterAggregatorTest.java index 5888b7d13dde..d914e71ccac5 100644 --- a/extensions-core/druid-bloom-filter/src/test/java/org/apache/druid/query/aggregation/bloom/BloomFilterAggregatorTest.java +++ b/extensions-core/druid-bloom-filter/src/test/java/org/apache/druid/query/aggregation/bloom/BloomFilterAggregatorTest.java @@ -158,7 +158,7 @@ private static void createStringFilter(List values, BloomKFilter filte private static List dimensionValues(Object... values) { return Lists.transform( - Lists.newArrayList(values), new Function() + Lists.newArrayList(values), new Function<>() { @Nullable @Override diff --git a/extensions-core/druid-catalog/src/main/java/org/apache/druid/catalog/storage/sql/SQLCatalogManager.java b/extensions-core/druid-catalog/src/main/java/org/apache/druid/catalog/storage/sql/SQLCatalogManager.java index fa24bd825613..a2300fc7c404 100644 --- a/extensions-core/druid-catalog/src/main/java/org/apache/druid/catalog/storage/sql/SQLCatalogManager.java +++ b/extensions-core/druid-catalog/src/main/java/org/apache/druid/catalog/storage/sql/SQLCatalogManager.java @@ -142,7 +142,7 @@ public long create(TableMetadata table) throws DuplicateKeyException { try { return dbi.withHandle( - new HandleCallback() + new HandleCallback<>() { @Override public Long withHandle(Handle handle) throws DuplicateKeyException @@ -197,7 +197,7 @@ public TableMetadata read(TableId id) throws NotFoundException { try { return dbi.withHandle( - new HandleCallback() + new HandleCallback<>() { @Override public TableMetadata withHandle(Handle handle) throws NotFoundException @@ -248,7 +248,7 @@ public long replace(TableMetadata table) throws NotFoundException { try { final TableMetadata revised = dbi.withHandle( - new HandleCallback() + new HandleCallback<>() { @Override public TableMetadata withHandle(Handle handle) throws NotFoundException @@ -293,7 +293,7 @@ public long update(TableMetadata table, long oldVersion) throws NotFoundExceptio { try { final TableMetadata revised = dbi.withHandle( - new HandleCallback() + new HandleCallback<>() { @Override public TableMetadata withHandle(Handle handle) throws NotFoundException @@ -354,7 +354,7 @@ public long updateProperties( { try { final TableMetadata result = dbi.withHandle( - new HandleCallback() + new HandleCallback<>() { @Override public TableMetadata withHandle(Handle handle) throws CatalogException @@ -446,7 +446,7 @@ public long updateColumns( { try { final TableMetadata result = dbi.withHandle( - new HandleCallback() + new HandleCallback<>() { @Override public TableMetadata withHandle(Handle handle) throws CatalogException @@ -527,7 +527,7 @@ public TableMetadata withHandle(Handle handle) throws CatalogException public long markDeleting(TableId id) { return dbi.withHandle( - new HandleCallback() + new HandleCallback<>() { @Override public Long withHandle(Handle handle) @@ -593,7 +593,7 @@ public Void withHandle(Handle handle) throws NotFoundException public List allTablePaths() { return dbi.withHandle( - new HandleCallback>() + new HandleCallback<>() { @Override public List withHandle(Handle handle) @@ -621,7 +621,7 @@ public List withHandle(Handle handle) public List tableNamesInSchema(String dbSchema) { return dbi.withHandle( - new HandleCallback>() + new HandleCallback<>() { @Override public List withHandle(Handle handle) @@ -650,7 +650,7 @@ public List withHandle(Handle handle) public List tablesInSchema(String dbSchema) { return dbi.withHandle( - new HandleCallback>() + new HandleCallback<>() { @Override public List withHandle(Handle handle) @@ -762,8 +762,7 @@ private static TableSpec tableSpecFromBytes( ); } - private static final TypeReference> PROPERTIES_TYPE_REF = - new TypeReference>() { }; + private static final TypeReference> PROPERTIES_TYPE_REF = new TypeReference<>() {}; private static Map propertiesFromBytes( final ObjectMapper jsonMapper, @@ -773,8 +772,7 @@ private static Map propertiesFromBytes( return fromBytes(jsonMapper, properties, PROPERTIES_TYPE_REF); } - private static final TypeReference> COLUMNS_TYPE_REF = - new TypeReference>() { }; + private static final TypeReference> COLUMNS_TYPE_REF = new TypeReference<>() {}; private static List columnsFromBytes( final ObjectMapper jsonMapper, diff --git a/extensions-core/druid-catalog/src/main/java/org/apache/druid/catalog/sync/CatalogClient.java b/extensions-core/druid-catalog/src/main/java/org/apache/druid/catalog/sync/CatalogClient.java index 5426f001a34b..21c198480908 100644 --- a/extensions-core/druid-catalog/src/main/java/org/apache/druid/catalog/sync/CatalogClient.java +++ b/extensions-core/druid-catalog/src/main/java/org/apache/druid/catalog/sync/CatalogClient.java @@ -59,13 +59,9 @@ public class CatalogClient implements CatalogSource { public static final String SCHEMA_SYNC_PATH = CatalogResource.ROOT_PATH + CatalogResource.SCHEMA_SYNC; public static final String TABLE_SYNC_PATH = CatalogResource.ROOT_PATH + CatalogResource.TABLE_SYNC; - private static final TypeReference> LIST_OF_TABLE_METADATA_TYPE = new TypeReference>() - { - }; + private static final TypeReference> LIST_OF_TABLE_METADATA_TYPE = new TypeReference<>() {}; // Not strictly needed as a TypeReference, but doing so makes the code simpler. - private static final TypeReference TABLE_METADATA_TYPE = new TypeReference() - { - }; + private static final TypeReference TABLE_METADATA_TYPE = new TypeReference<>() {}; private final DruidLeaderClient coordClient; private final ObjectMapper smileMapper; diff --git a/extensions-core/druid-kerberos/src/main/java/org/apache/druid/security/kerberos/DruidKerberosAuthenticationHandler.java b/extensions-core/druid-kerberos/src/main/java/org/apache/druid/security/kerberos/DruidKerberosAuthenticationHandler.java index add376761a92..16eb3b28e6b9 100644 --- a/extensions-core/druid-kerberos/src/main/java/org/apache/druid/security/kerberos/DruidKerberosAuthenticationHandler.java +++ b/extensions-core/druid-kerberos/src/main/java/org/apache/druid/security/kerberos/DruidKerberosAuthenticationHandler.java @@ -53,7 +53,7 @@ public class DruidKerberosAuthenticationHandler extends KerberosAuthenticationHa private String keytab; private GSSManager gssManager; private Subject serverSubject = new Subject(); - private List loginContexts = new ArrayList(); + private List loginContexts = new ArrayList<>(); @Override public void destroy() @@ -120,7 +120,7 @@ public void init(Properties config) throws ServletException loginContexts.add(loginContext); } try { - gssManager = Subject.doAs(serverSubject, new PrivilegedExceptionAction() + gssManager = Subject.doAs(serverSubject, new PrivilegedExceptionAction<>() { @Override @@ -157,7 +157,7 @@ public AuthenticationToken authenticate(HttpServletRequest request, final HttpSe final byte[] clientToken = StringUtils.decodeBase64String(authorization); final String serverName = request.getServerName(); try { - token = Subject.doAs(serverSubject, new PrivilegedExceptionAction() + token = Subject.doAs(serverSubject, new PrivilegedExceptionAction<>() { @Override diff --git a/extensions-core/druid-kerberos/src/main/java/org/apache/druid/security/kerberos/KerberosAuthenticator.java b/extensions-core/druid-kerberos/src/main/java/org/apache/druid/security/kerberos/KerberosAuthenticator.java index a58c7997aa48..3e0e04e215e7 100644 --- a/extensions-core/druid-kerberos/src/main/java/org/apache/druid/security/kerberos/KerberosAuthenticator.java +++ b/extensions-core/druid-kerberos/src/main/java/org/apache/druid/security/kerberos/KerberosAuthenticator.java @@ -376,7 +376,7 @@ public Class getFilterClass() @Override public Map getInitParameters() { - Map params = new HashMap(); + Map params = new HashMap<>(); params.put("kerberos.principal", serverPrincipal); params.put("kerberos.keytab", serverKeytab); params.put(AuthenticationFilter.AUTH_TYPE, DruidKerberosAuthenticationHandler.class.getName()); @@ -447,7 +447,7 @@ public DruidKerberosConfiguration(String keytab, String principal) @Override public AppConfigurationEntry[] getAppConfigurationEntry(String name) { - Map options = new HashMap(); + Map options = new HashMap<>(); if (System.getProperty("java.vendor").contains("IBM")) { options.put( "useKeytab", @@ -508,9 +508,9 @@ private void initializeKerberosLogin() throws ServletException throw new ServletException("Keytab does not exist: " + keytab); } - Set principals = new HashSet(); + Set principals = new HashSet<>(); principals.add(new KerberosPrincipal(serverPrincipal)); - Subject subject = new Subject(false, principals, new HashSet(), new HashSet()); + Subject subject = new Subject(false, principals, new HashSet<>(), new HashSet<>()); DruidKerberosConfiguration kerberosConfiguration = new DruidKerberosConfiguration(keytab, serverPrincipal); diff --git a/extensions-core/druid-kerberos/src/main/java/org/apache/druid/security/kerberos/KerberosHttpClient.java b/extensions-core/druid-kerberos/src/main/java/org/apache/druid/security/kerberos/KerberosHttpClient.java index eee32ac44e24..da1cc598cabd 100644 --- a/extensions-core/druid-kerberos/src/main/java/org/apache/druid/security/kerberos/KerberosHttpClient.java +++ b/extensions-core/druid-kerberos/src/main/java/org/apache/druid/security/kerberos/KerberosHttpClient.java @@ -99,7 +99,7 @@ private void inner_go( ); DruidKerberosUtil.authenticateIfRequired(internalClientPrincipal, internalClientKeytab); UserGroupInformation currentUser = UserGroupInformation.getCurrentUser(); - String challenge = currentUser.doAs(new PrivilegedExceptionAction() + String challenge = currentUser.doAs(new PrivilegedExceptionAction<>() { @Override public String run() throws Exception @@ -122,7 +122,7 @@ public String run() throws Exception duration ); - Futures.addCallback(internalFuture, new FutureCallback>() + Futures.addCallback(internalFuture, new FutureCallback<>() { @Override public void onSuccess(RetryResponseHolder result) diff --git a/extensions-core/druid-kerberos/src/main/java/org/apache/druid/security/kerberos/ResponseCookieHandler.java b/extensions-core/druid-kerberos/src/main/java/org/apache/druid/security/kerberos/ResponseCookieHandler.java index 50973447b57d..ece3bc69906a 100644 --- a/extensions-core/druid-kerberos/src/main/java/org/apache/druid/security/kerberos/ResponseCookieHandler.java +++ b/extensions-core/druid-kerberos/src/main/java/org/apache/druid/security/kerberos/ResponseCookieHandler.java @@ -53,7 +53,7 @@ public ClientResponse handleResponse(HttpResponse httpResponse, Tr { try { final HttpHeaders headers = httpResponse.headers(); - manager.put(uri, Maps.asMap(headers.names(), new Function>() + manager.put(uri, Maps.asMap(headers.names(), new Function<>() { @Override public List apply(String input) diff --git a/extensions-core/druid-kerberos/src/main/java/org/apache/druid/security/kerberos/RetryResponseHolder.java b/extensions-core/druid-kerberos/src/main/java/org/apache/druid/security/kerberos/RetryResponseHolder.java index acfb747914ee..c8721f1d3400 100644 --- a/extensions-core/druid-kerberos/src/main/java/org/apache/druid/security/kerberos/RetryResponseHolder.java +++ b/extensions-core/druid-kerberos/src/main/java/org/apache/druid/security/kerberos/RetryResponseHolder.java @@ -32,7 +32,7 @@ public RetryResponseHolder(boolean shouldRetry, T obj) public static RetryResponseHolder retry() { - return new RetryResponseHolder(true, null); + return new RetryResponseHolder<>(true, null); } public boolean shouldRetry() diff --git a/extensions-core/druid-pac4j/src/main/java/org/apache/druid/security/pac4j/OIDCConfig.java b/extensions-core/druid-pac4j/src/main/java/org/apache/druid/security/pac4j/OIDCConfig.java index 376181416556..50b04455dbc5 100644 --- a/extensions-core/druid-pac4j/src/main/java/org/apache/druid/security/pac4j/OIDCConfig.java +++ b/extensions-core/druid-pac4j/src/main/java/org/apache/druid/security/pac4j/OIDCConfig.java @@ -28,7 +28,7 @@ public class OIDCConfig { - private final String DEFAULT_SCOPE = "name"; + private static final String DEFAULT_SCOPE = "name"; @JsonProperty private final String clientID; diff --git a/extensions-core/ec2-extensions/src/main/java/org/apache/druid/indexing/overlord/autoscaling/ec2/EC2AutoScaler.java b/extensions-core/ec2-extensions/src/main/java/org/apache/druid/indexing/overlord/autoscaling/ec2/EC2AutoScaler.java index 56ffc6b74efc..7d996297b12e 100644 --- a/extensions-core/ec2-extensions/src/main/java/org/apache/druid/indexing/overlord/autoscaling/ec2/EC2AutoScaler.java +++ b/extensions-core/ec2-extensions/src/main/java/org/apache/druid/indexing/overlord/autoscaling/ec2/EC2AutoScaler.java @@ -152,7 +152,7 @@ public AutoScalingData provision() final List instanceIds = Lists.transform( result.getReservation().getInstances(), - new Function() + new Function<>() { @Override public String apply(Instance input) @@ -167,7 +167,7 @@ public String apply(Instance input) return new AutoScalingData( Lists.transform( result.getReservation().getInstances(), - new Function() + new Function<>() { @Override public String apply(Instance input) @@ -208,7 +208,7 @@ public AutoScalingData terminate(List ips) return terminateWithIds( Lists.transform( instances, - new Function() + new Function<>() { @Override public String apply(Instance input) diff --git a/extensions-core/google-extensions/pom.xml b/extensions-core/google-extensions/pom.xml index 39f89a2e5a53..e83a85f55535 100644 --- a/extensions-core/google-extensions/pom.xml +++ b/extensions-core/google-extensions/pom.xml @@ -82,11 +82,6 @@ jackson-core provided - - com.google.inject.extensions - guice-multibindings - provided - com.google.guava guava diff --git a/extensions-core/google-extensions/src/main/java/org/apache/druid/storage/google/GoogleDataSegmentPuller.java b/extensions-core/google-extensions/src/main/java/org/apache/druid/storage/google/GoogleDataSegmentPuller.java index 61595264fc67..8e155f6a4ce8 100644 --- a/extensions-core/google-extensions/src/main/java/org/apache/druid/storage/google/GoogleDataSegmentPuller.java +++ b/extensions-core/google-extensions/src/main/java/org/apache/druid/storage/google/GoogleDataSegmentPuller.java @@ -96,7 +96,7 @@ public String getVersion(URI uri) throws IOException @Override public Predicate shouldRetryPredicate() { - return new Predicate() + return new Predicate<>() { @Override public boolean apply(Throwable e) diff --git a/extensions-core/google-extensions/src/main/java/org/apache/druid/storage/google/output/GoogleStorageConnector.java b/extensions-core/google-extensions/src/main/java/org/apache/druid/storage/google/output/GoogleStorageConnector.java index 0d147cce25a3..69b3e28c8060 100644 --- a/extensions-core/google-extensions/src/main/java/org/apache/druid/storage/google/output/GoogleStorageConnector.java +++ b/extensions-core/google-extensions/src/main/java/org/apache/druid/storage/google/output/GoogleStorageConnector.java @@ -199,7 +199,7 @@ public ChunkingStorageConnectorParameters buildInputParams(Str config.getBucket(), objectPath(path) ))); - builder.objectOpenFunction(new ObjectOpenFunction() + builder.objectOpenFunction(new ObjectOpenFunction<>() { @Override public InputStream open(GoogleInputRange googleInputRange) throws IOException diff --git a/extensions-core/hdfs-storage/pom.xml b/extensions-core/hdfs-storage/pom.xml index ac8f7878e5d6..881ba1884716 100644 --- a/extensions-core/hdfs-storage/pom.xml +++ b/extensions-core/hdfs-storage/pom.xml @@ -88,11 +88,7 @@ jackson-core provided - - com.google.inject.extensions - guice-multibindings - provided - + org.apache.commons commons-lang3 diff --git a/extensions-core/hdfs-storage/src/main/java/org/apache/druid/storage/hdfs/HdfsDataSegmentPuller.java b/extensions-core/hdfs-storage/src/main/java/org/apache/druid/storage/hdfs/HdfsDataSegmentPuller.java index 1c0adbd133c3..5dc6d810cbcc 100644 --- a/extensions-core/hdfs-storage/src/main/java/org/apache/druid/storage/hdfs/HdfsDataSegmentPuller.java +++ b/extensions-core/hdfs-storage/src/main/java/org/apache/druid/storage/hdfs/HdfsDataSegmentPuller.java @@ -56,7 +56,7 @@ public class HdfsDataSegmentPuller implements URIDataPuller { public static final int DEFAULT_RETRY_COUNT = 3; - public static final Predicate RETRY_PREDICATE = new Predicate() + public static final Predicate RETRY_PREDICATE = new Predicate<>() { @Override public boolean apply(Throwable input) diff --git a/extensions-core/hdfs-storage/src/test/java/org/apache/druid/storage/hdfs/HdfsDataSegmentPusherTest.java b/extensions-core/hdfs-storage/src/test/java/org/apache/druid/storage/hdfs/HdfsDataSegmentPusherTest.java index b26c2858f4b9..7bdf8c06980f 100644 --- a/extensions-core/hdfs-storage/src/test/java/org/apache/druid/storage/hdfs/HdfsDataSegmentPusherTest.java +++ b/extensions-core/hdfs-storage/src/test/java/org/apache/druid/storage/hdfs/HdfsDataSegmentPusherTest.java @@ -397,7 +397,7 @@ public static class TestModule extends SimpleModule addSerializer(NumberedShardSpec.class, ToStringSerializer.instance); addDeserializer( Interval.class, - new StdDeserializer(Interval.class) + new StdDeserializer<>(Interval.class) { @Override public Interval deserialize(JsonParser jsonParser, DeserializationContext deserializationContext) diff --git a/extensions-core/histogram/src/main/java/org/apache/druid/query/aggregation/histogram/ApproximateHistogram.java b/extensions-core/histogram/src/main/java/org/apache/druid/query/aggregation/histogram/ApproximateHistogram.java index fa3772518b3e..11236b8d25fe 100644 --- a/extensions-core/histogram/src/main/java/org/apache/druid/query/aggregation/histogram/ApproximateHistogram.java +++ b/extensions-core/histogram/src/main/java/org/apache/druid/query/aggregation/histogram/ApproximateHistogram.java @@ -41,8 +41,8 @@ public class ApproximateHistogram // max size of the histogram (number of bincount/position pairs) int size; - public float[] positions; - public long[] bins; + protected float[] positions; + protected long[] bins; // used bincount int binCount; @@ -1659,7 +1659,7 @@ public Histogram toHistogram(final float bucketSize, final float offset) final float cutoff = 0.1f; - final ArrayList breaks = new ArrayList(); + final ArrayList breaks = new ArrayList<>(); // to deal with left inclusivity when the min is the same as a break final float bottomBreak = minFloor - bucketSize; diff --git a/extensions-core/histogram/src/main/java/org/apache/druid/query/aggregation/histogram/ApproximateHistogramFoldingSerde.java b/extensions-core/histogram/src/main/java/org/apache/druid/query/aggregation/histogram/ApproximateHistogramFoldingSerde.java index dd35874b8e27..31784f0d744a 100644 --- a/extensions-core/histogram/src/main/java/org/apache/druid/query/aggregation/histogram/ApproximateHistogramFoldingSerde.java +++ b/extensions-core/histogram/src/main/java/org/apache/druid/query/aggregation/histogram/ApproximateHistogramFoldingSerde.java @@ -82,7 +82,7 @@ public ApproximateHistogram extractValue(InputRow inputRow, String metricName) @Override public ObjectStrategy getObjectStrategy() { - return new ObjectStrategy() + return new ObjectStrategy<>() { @Override public Class getClazz() diff --git a/extensions-core/histogram/src/test/java/org/apache/druid/query/aggregation/histogram/ApproximateHistogramPostAggregatorTest.java b/extensions-core/histogram/src/test/java/org/apache/druid/query/aggregation/histogram/ApproximateHistogramPostAggregatorTest.java index bcdc7d46e98c..232979f677ea 100644 --- a/extensions-core/histogram/src/test/java/org/apache/druid/query/aggregation/histogram/ApproximateHistogramPostAggregatorTest.java +++ b/extensions-core/histogram/src/test/java/org/apache/druid/query/aggregation/histogram/ApproximateHistogramPostAggregatorTest.java @@ -60,7 +60,7 @@ public void testApproxHistogramCompute() selector.increment(); } - Map metricValues = new HashMap(); + Map metricValues = new HashMap<>(); metricValues.put("price", agg.get()); ApproximateHistogramPostAggregator approximateHistogramPostAggregator = new EqualBucketsPostAggregator( diff --git a/extensions-core/histogram/src/test/java/org/apache/druid/query/aggregation/histogram/ApproximateHistogramTopNQueryTest.java b/extensions-core/histogram/src/test/java/org/apache/druid/query/aggregation/histogram/ApproximateHistogramTopNQueryTest.java index ae6c634ad0bc..531b530328b7 100644 --- a/extensions-core/histogram/src/test/java/org/apache/druid/query/aggregation/histogram/ApproximateHistogramTopNQueryTest.java +++ b/extensions-core/histogram/src/test/java/org/apache/druid/query/aggregation/histogram/ApproximateHistogramTopNQueryTest.java @@ -114,7 +114,7 @@ public void testTopNWithApproximateHistogramAgg() .build(); List> expectedResults = Collections.singletonList( - new Result( + new Result<>( DateTimes.of("2011-01-12T00:00:00.000Z"), TopNResultValue.create( Arrays.>asList( diff --git a/extensions-core/histogram/src/test/java/org/apache/druid/query/aggregation/histogram/FixedBucketsHistogramTopNQueryTest.java b/extensions-core/histogram/src/test/java/org/apache/druid/query/aggregation/histogram/FixedBucketsHistogramTopNQueryTest.java index cead791167d3..474cd9759dac 100644 --- a/extensions-core/histogram/src/test/java/org/apache/druid/query/aggregation/histogram/FixedBucketsHistogramTopNQueryTest.java +++ b/extensions-core/histogram/src/test/java/org/apache/druid/query/aggregation/histogram/FixedBucketsHistogramTopNQueryTest.java @@ -113,7 +113,7 @@ public void testTopNWithFixedHistogramAgg() .build(); List> expectedResults = Collections.singletonList( - new Result( + new Result<>( DateTimes.of("2011-01-12T00:00:00.000Z"), TopNResultValue.create( Arrays.>asList( diff --git a/extensions-core/kafka-indexing-service/src/main/java/org/apache/druid/data/input/kafkainput/KafkaInputReader.java b/extensions-core/kafka-indexing-service/src/main/java/org/apache/druid/data/input/kafkainput/KafkaInputReader.java index 1ce992d7cc37..b0964dfe5467 100644 --- a/extensions-core/kafka-indexing-service/src/main/java/org/apache/druid/data/input/kafkainput/KafkaInputReader.java +++ b/extensions-core/kafka-indexing-service/src/main/java/org/apache/druid/data/input/kafkainput/KafkaInputReader.java @@ -289,7 +289,7 @@ private static Map buildBlendedEventMap( final Set keySet = new HashSet<>(fallback.keySet()); keySet.addAll(rowDimensions); - return new AbstractMap() + return new AbstractMap<>() { @Override public Object get(Object key) diff --git a/extensions-core/kafka-indexing-service/src/main/java/org/apache/druid/indexing/kafka/IncrementalPublishingKafkaIndexTaskRunner.java b/extensions-core/kafka-indexing-service/src/main/java/org/apache/druid/indexing/kafka/IncrementalPublishingKafkaIndexTaskRunner.java index 5756d6a3fae5..b16f825308fc 100644 --- a/extensions-core/kafka-indexing-service/src/main/java/org/apache/druid/indexing/kafka/IncrementalPublishingKafkaIndexTaskRunner.java +++ b/extensions-core/kafka-indexing-service/src/main/java/org/apache/druid/indexing/kafka/IncrementalPublishingKafkaIndexTaskRunner.java @@ -218,9 +218,7 @@ protected boolean isEndOfShard(Long seqNum) @Override public TypeReference>> getSequenceMetadataTypeReference() { - return new TypeReference>>() - { - }; + return new TypeReference<>() {}; } @Nullable @@ -234,9 +232,7 @@ protected TreeMap> getCheckPointsFromCon log.debug("Got checkpoints from task context[%s].", checkpointsString); return toolbox.getJsonMapper().readValue( checkpointsString, - new TypeReference>>() - { - } + new TypeReference<>() {} ); } else { return null; diff --git a/extensions-core/kafka-indexing-service/src/main/java/org/apache/druid/indexing/kafka/supervisor/KafkaSupervisor.java b/extensions-core/kafka-indexing-service/src/main/java/org/apache/druid/indexing/kafka/supervisor/KafkaSupervisor.java index 2618c22495ea..f449c158789a 100644 --- a/extensions-core/kafka-indexing-service/src/main/java/org/apache/druid/indexing/kafka/supervisor/KafkaSupervisor.java +++ b/extensions-core/kafka-indexing-service/src/main/java/org/apache/druid/indexing/kafka/supervisor/KafkaSupervisor.java @@ -85,9 +85,7 @@ public class KafkaSupervisor extends SeekableStreamSupervisor { public static final TypeReference>> CHECKPOINTS_TYPE_REF = - new TypeReference>>() - { - }; + new TypeReference<>() {}; private static final EmittingLogger log = new EmittingLogger(KafkaSupervisor.class); private static final Long NOT_SET = -1L; diff --git a/extensions-core/kafka-indexing-service/src/test/java/org/apache/druid/data/input/kafkainput/KafkaStringHeaderFormatTest.java b/extensions-core/kafka-indexing-service/src/test/java/org/apache/druid/data/input/kafkainput/KafkaStringHeaderFormatTest.java index ab634b3d6715..913c1aca1ad9 100644 --- a/extensions-core/kafka-indexing-service/src/test/java/org/apache/druid/data/input/kafkainput/KafkaStringHeaderFormatTest.java +++ b/extensions-core/kafka-indexing-service/src/test/java/org/apache/druid/data/input/kafkainput/KafkaStringHeaderFormatTest.java @@ -100,7 +100,7 @@ public void testDefaultHeaderFormat() { String headerLabelPrefix = "test.kafka.header."; Headers headers = new RecordHeaders(SAMPLE_HEADERS); - inputEntity = new KafkaRecordEntity(new ConsumerRecord( + inputEntity = new KafkaRecordEntity(new ConsumerRecord<>( "sample", 0, 0, timestamp, null, null, 0, 0, null, "sampleValue".getBytes(StandardCharsets.UTF_8), headers @@ -151,7 +151,7 @@ public byte[] value() String headerLabelPrefix = "test.kafka.header."; Headers headers = new RecordHeaders(header); - inputEntity = new KafkaRecordEntity(new ConsumerRecord( + inputEntity = new KafkaRecordEntity(new ConsumerRecord<>( "sample", 0, 0, timestamp, null, null, 0, 0, null, "sampleValue".getBytes(StandardCharsets.UTF_8), headers @@ -203,7 +203,7 @@ public byte[] value() String headerLabelPrefix = "test.kafka.header."; Headers headers = new RecordHeaders(header); - inputEntity = new KafkaRecordEntity(new ConsumerRecord( + inputEntity = new KafkaRecordEntity(new ConsumerRecord<>( "sample", 0, 0, timestamp, null, null, 0, 0, null, "sampleValue".getBytes(StandardCharsets.UTF_8), headers diff --git a/extensions-core/kafka-indexing-service/src/test/java/org/apache/druid/indexing/kafka/KafkaIndexTaskTest.java b/extensions-core/kafka-indexing-service/src/test/java/org/apache/druid/indexing/kafka/KafkaIndexTaskTest.java index 3bae10a1969b..a0eb4e942aa3 100644 --- a/extensions-core/kafka-indexing-service/src/test/java/org/apache/druid/indexing/kafka/KafkaIndexTaskTest.java +++ b/extensions-core/kafka-indexing-service/src/test/java/org/apache/druid/indexing/kafka/KafkaIndexTaskTest.java @@ -2270,6 +2270,8 @@ public void testRunWithPauseAndResume() throws Exception ) ); + Assert.assertEquals(Status.NOT_STARTED.toString(), task.getCurrentRunnerStatus()); + final ListenableFuture future = runTask(task); // Insert some data, but not enough for the task to finish @@ -2280,14 +2282,15 @@ public void testRunWithPauseAndResume() throws Exception } Assert.assertEquals(2, countEvents(task)); Assert.assertEquals(Status.READING, task.getRunner().getStatus()); + Assert.assertEquals(Status.READING.toString(), task.getCurrentRunnerStatus()); + Map currentOffsets = OBJECT_MAPPER.readValue( task.getRunner().pause().getEntity().toString(), - new TypeReference>() - { - } + new TypeReference<>() {} ); Assert.assertEquals(Status.PAUSED, task.getRunner().getStatus()); + Assert.assertEquals(Status.PAUSED.toString(), task.getCurrentRunnerStatus()); // Insert remaining data insertData(Iterables.skip(records, 4)); @@ -2305,6 +2308,7 @@ public void testRunWithPauseAndResume() throws Exception Assert.assertEquals(TaskState.SUCCESS, future.get().getStatusCode()); Assert.assertEquals(task.getRunner().getEndOffsets(), task.getRunner().getCurrentOffsets()); + Assert.assertEquals(Status.PUBLISHING.toString(), task.getCurrentRunnerStatus()); verifyTaskMetrics(task, RowMeters.with().bytes(getTotalSizeOfRecords(2, 5)).totalProcessed(3)); // Check published metadata and segments in deep storage diff --git a/extensions-core/kafka-indexing-service/src/test/java/org/apache/druid/indexing/kafka/KafkaSeekableStreamEndSequenceNumbersTest.java b/extensions-core/kafka-indexing-service/src/test/java/org/apache/druid/indexing/kafka/KafkaSeekableStreamEndSequenceNumbersTest.java index e3b3cef6cb20..d8aa85369359 100644 --- a/extensions-core/kafka-indexing-service/src/test/java/org/apache/druid/indexing/kafka/KafkaSeekableStreamEndSequenceNumbersTest.java +++ b/extensions-core/kafka-indexing-service/src/test/java/org/apache/druid/indexing/kafka/KafkaSeekableStreamEndSequenceNumbersTest.java @@ -61,7 +61,7 @@ public void testSerde() throws Exception // Check round-trip. final SeekableStreamEndSequenceNumbers partitions2 = OBJECT_MAPPER.readValue( serializedString, - new TypeReference>() {} + new TypeReference<>() {} ); Assert.assertEquals( diff --git a/extensions-core/kafka-indexing-service/src/test/java/org/apache/druid/indexing/kafka/KafkaSeekableStreamStartSequenceNumbersTest.java b/extensions-core/kafka-indexing-service/src/test/java/org/apache/druid/indexing/kafka/KafkaSeekableStreamStartSequenceNumbersTest.java index 07eaced75559..5ed51f092a84 100644 --- a/extensions-core/kafka-indexing-service/src/test/java/org/apache/druid/indexing/kafka/KafkaSeekableStreamStartSequenceNumbersTest.java +++ b/extensions-core/kafka-indexing-service/src/test/java/org/apache/druid/indexing/kafka/KafkaSeekableStreamStartSequenceNumbersTest.java @@ -66,7 +66,7 @@ public void testSerde() throws Exception // Check round-trip. final SeekableStreamStartSequenceNumbers partitions2 = OBJECT_MAPPER.readValue( serializedString, - new TypeReference>() {} + new TypeReference<>() {} ); Assert.assertEquals( diff --git a/extensions-core/kafka-indexing-service/src/test/java/org/apache/druid/indexing/kafka/supervisor/KafkaSupervisorTest.java b/extensions-core/kafka-indexing-service/src/test/java/org/apache/druid/indexing/kafka/supervisor/KafkaSupervisorTest.java index 8f2f3167ecd9..40a97f1e1ada 100644 --- a/extensions-core/kafka-indexing-service/src/test/java/org/apache/druid/indexing/kafka/supervisor/KafkaSupervisorTest.java +++ b/extensions-core/kafka-indexing-service/src/test/java/org/apache/druid/indexing/kafka/supervisor/KafkaSupervisorTest.java @@ -2666,7 +2666,7 @@ public void testSupervisorIsIdleIfStreamInactiveWhenNoActiveTasks() throws Excep taskRunner.registerListener(EasyMock.anyObject(TaskRunnerListener.class), EasyMock.anyObject(Executor.class)); EasyMock.expect(indexerMetadataStorageCoordinator.retrieveDataSourceMetadata(DATASOURCE)).andReturn( new KafkaDataSourceMetadata( - new SeekableStreamEndSequenceNumbers( + new SeekableStreamEndSequenceNumbers<>( topic, singlePartitionMap(topic, 0, @@ -2724,7 +2724,7 @@ public void testSupervisorNotIdleIfStreamInactiveWhenSuspended() throws Exceptio taskRunner.registerListener(EasyMock.anyObject(TaskRunnerListener.class), EasyMock.anyObject(Executor.class)); EasyMock.expect(indexerMetadataStorageCoordinator.retrieveDataSourceMetadata(DATASOURCE)).andReturn( new KafkaDataSourceMetadata( - new SeekableStreamEndSequenceNumbers( + new SeekableStreamEndSequenceNumbers<>( topic, singlePartitionMap(topic, 0, @@ -2784,7 +2784,7 @@ public void testSupervisorIsIdleIfStreamInactiveWhenSuspended() throws Exception taskRunner.registerListener(EasyMock.anyObject(TaskRunnerListener.class), EasyMock.anyObject(Executor.class)); EasyMock.expect(indexerMetadataStorageCoordinator.retrieveDataSourceMetadata(DATASOURCE)).andReturn( new KafkaDataSourceMetadata( - new SeekableStreamEndSequenceNumbers( + new SeekableStreamEndSequenceNumbers<>( topic, singlePartitionMap(topic, 0, diff --git a/extensions-core/kinesis-indexing-service/src/main/java/org/apache/druid/data/input/kinesis/KinesisInputReader.java b/extensions-core/kinesis-indexing-service/src/main/java/org/apache/druid/data/input/kinesis/KinesisInputReader.java index d0c30280a2b7..b5633c1c3c7e 100644 --- a/extensions-core/kinesis-indexing-service/src/main/java/org/apache/druid/data/input/kinesis/KinesisInputReader.java +++ b/extensions-core/kinesis-indexing-service/src/main/java/org/apache/druid/data/input/kinesis/KinesisInputReader.java @@ -204,7 +204,7 @@ private Map buildBlendedEventMap( final Set keySet = new HashSet<>(fallback.keySet()); keySet.addAll(rowDimensions); - return new AbstractMap() + return new AbstractMap<>() { @Override public Object get(Object key) diff --git a/extensions-core/kinesis-indexing-service/src/main/java/org/apache/druid/indexing/kinesis/KinesisIndexTaskRunner.java b/extensions-core/kinesis-indexing-service/src/main/java/org/apache/druid/indexing/kinesis/KinesisIndexTaskRunner.java index 72e61635912c..dfca894d2819 100644 --- a/extensions-core/kinesis-indexing-service/src/main/java/org/apache/druid/indexing/kinesis/KinesisIndexTaskRunner.java +++ b/extensions-core/kinesis-indexing-service/src/main/java/org/apache/druid/indexing/kinesis/KinesisIndexTaskRunner.java @@ -181,9 +181,7 @@ protected boolean isEndOfShard(String seqNum) @Override public TypeReference>> getSequenceMetadataTypeReference() { - return new TypeReference>>() - { - }; + return new TypeReference<>() {}; } @Nullable @@ -197,9 +195,7 @@ protected TreeMap> getCheckPointsFromContext( log.debug("Got checkpoints from task context[%s]", checkpointsString); return toolbox.getJsonMapper().readValue( checkpointsString, - new TypeReference>>() - { - } + new TypeReference<>() {} ); } else { return null; diff --git a/extensions-core/kinesis-indexing-service/src/main/java/org/apache/druid/indexing/kinesis/supervisor/KinesisSupervisor.java b/extensions-core/kinesis-indexing-service/src/main/java/org/apache/druid/indexing/kinesis/supervisor/KinesisSupervisor.java index 2391b1265bab..db29d0dae53d 100644 --- a/extensions-core/kinesis-indexing-service/src/main/java/org/apache/druid/indexing/kinesis/supervisor/KinesisSupervisor.java +++ b/extensions-core/kinesis-indexing-service/src/main/java/org/apache/druid/indexing/kinesis/supervisor/KinesisSupervisor.java @@ -79,9 +79,7 @@ public class KinesisSupervisor extends SeekableStreamSupervisor>> CHECKPOINTS_TYPE_REF = - new TypeReference>>() - { - }; + new TypeReference<>() {}; public static final String OFFSET_NOT_SET = "-1"; private final KinesisSupervisorSpec spec; diff --git a/extensions-core/kinesis-indexing-service/src/test/java/org/apache/druid/indexing/kinesis/supervisor/KinesisSupervisorTest.java b/extensions-core/kinesis-indexing-service/src/test/java/org/apache/druid/indexing/kinesis/supervisor/KinesisSupervisorTest.java index dc10e348138a..1fc1da973859 100644 --- a/extensions-core/kinesis-indexing-service/src/test/java/org/apache/druid/indexing/kinesis/supervisor/KinesisSupervisorTest.java +++ b/extensions-core/kinesis-indexing-service/src/test/java/org/apache/druid/indexing/kinesis/supervisor/KinesisSupervisorTest.java @@ -4311,7 +4311,7 @@ private List testShardSplitPhaseTwo(List phaseOneTasks) throws Excep // first task ran, its shard 0 has reached EOS EasyMock.expect(indexerMetadataStorageCoordinator.retrieveDataSourceMetadata(DATASOURCE)).andReturn( new KinesisDataSourceMetadata( - new SeekableStreamEndSequenceNumbers( + new SeekableStreamEndSequenceNumbers<>( STREAM, ImmutableMap.of(SHARD_ID0, KinesisSequenceNumber.END_OF_SHARD_MARKER) ) @@ -4468,7 +4468,7 @@ private void testShardSplitPhaseThree(List phaseTwoTasks) throws Exception // second set of tasks ran, shard 0 has expired, but shard 1 and 2 have data EasyMock.expect(indexerMetadataStorageCoordinator.retrieveDataSourceMetadata(DATASOURCE)).andReturn( new KinesisDataSourceMetadata( - new SeekableStreamEndSequenceNumbers( + new SeekableStreamEndSequenceNumbers<>( STREAM, ImmutableMap.of( SHARD_ID0, KinesisSequenceNumber.END_OF_SHARD_MARKER, @@ -4483,7 +4483,7 @@ private void testShardSplitPhaseThree(List phaseTwoTasks) throws Exception indexerMetadataStorageCoordinator.resetDataSourceMetadata( DATASOURCE, new KinesisDataSourceMetadata( - new SeekableStreamEndSequenceNumbers( + new SeekableStreamEndSequenceNumbers<>( STREAM, ImmutableMap.of( SHARD_ID0, KinesisSequenceNumber.EXPIRED_MARKER, @@ -4768,7 +4768,7 @@ private List testShardMergePhaseTwo(List phaseOneTasks) throws Excep // first tasks ran, both shard 0 and shard 1 have reached EOS, merged into shard 2 EasyMock.expect(indexerMetadataStorageCoordinator.retrieveDataSourceMetadata(DATASOURCE)).andReturn( new KinesisDataSourceMetadata( - new SeekableStreamEndSequenceNumbers( + new SeekableStreamEndSequenceNumbers<>( STREAM, ImmutableMap.of( SHARD_ID0, KinesisSequenceNumber.END_OF_SHARD_MARKER, @@ -4909,7 +4909,7 @@ private void testShardMergePhaseThree(List phaseTwoTasks) throws Exception // second set of tasks ran, shard 0 has expired, but shard 1 and 2 have data EasyMock.expect(indexerMetadataStorageCoordinator.retrieveDataSourceMetadata(DATASOURCE)).andReturn( new KinesisDataSourceMetadata( - new SeekableStreamEndSequenceNumbers( + new SeekableStreamEndSequenceNumbers<>( STREAM, ImmutableMap.of( SHARD_ID0, KinesisSequenceNumber.END_OF_SHARD_MARKER, @@ -4924,7 +4924,7 @@ private void testShardMergePhaseThree(List phaseTwoTasks) throws Exception indexerMetadataStorageCoordinator.resetDataSourceMetadata( DATASOURCE, new KinesisDataSourceMetadata( - new SeekableStreamEndSequenceNumbers( + new SeekableStreamEndSequenceNumbers<>( STREAM, ImmutableMap.of( SHARD_ID0, KinesisSequenceNumber.EXPIRED_MARKER, diff --git a/extensions-core/kubernetes-extensions/pom.xml b/extensions-core/kubernetes-extensions/pom.xml index 6ef180795e25..e9ca2cae0e35 100644 --- a/extensions-core/kubernetes-extensions/pom.xml +++ b/extensions-core/kubernetes-extensions/pom.xml @@ -109,11 +109,6 @@ jackson-databind provided - - com.google.inject.extensions - guice-multibindings - provided - joda-time joda-time diff --git a/extensions-core/kubernetes-extensions/src/main/java/org/apache/druid/k8s/discovery/DefaultK8sApiClient.java b/extensions-core/kubernetes-extensions/src/main/java/org/apache/druid/k8s/discovery/DefaultK8sApiClient.java index 00ad6b76abb2..03a1b6914a85 100644 --- a/extensions-core/kubernetes-extensions/src/main/java/org/apache/druid/k8s/discovery/DefaultK8sApiClient.java +++ b/extensions-core/kubernetes-extensions/src/main/java/org/apache/druid/k8s/discovery/DefaultK8sApiClient.java @@ -145,7 +145,7 @@ public boolean hasNext() throws SocketTimeoutException LOGGER.debug("item of type " + item.type + " was NULL when watching nodeRole [%s]", nodeRole); } - obj = new Watch.Response( + obj = new Watch.Response<>( item.type, result ); diff --git a/extensions-core/kubernetes-extensions/src/test/java/org/apache/druid/k8s/discovery/K8sDruidNodeAnnouncerTest.java b/extensions-core/kubernetes-extensions/src/test/java/org/apache/druid/k8s/discovery/K8sDruidNodeAnnouncerTest.java index ffcaefb56177..fc0304009f73 100644 --- a/extensions-core/kubernetes-extensions/src/test/java/org/apache/druid/k8s/discovery/K8sDruidNodeAnnouncerTest.java +++ b/extensions-core/kubernetes-extensions/src/test/java/org/apache/druid/k8s/discovery/K8sDruidNodeAnnouncerTest.java @@ -67,9 +67,7 @@ public void testAnnounce() throws Exception List> actualPatchList = jsonMapper.readValue( patchArg.getValue(), - new TypeReference>>() - { - } + new TypeReference<>() {} ); List> expectedPatchList = Lists.newArrayList( @@ -115,9 +113,7 @@ public void testUnannounce() throws Exception List> actualPatchList = jsonMapper.readValue( patchArg.getValue(), - new TypeReference>>() - { - } + new TypeReference<>() {} ); List> expectedPatchList = Lists.newArrayList( diff --git a/extensions-core/kubernetes-extensions/src/test/java/org/apache/druid/k8s/discovery/K8sDruidNodeDiscoveryProviderTest.java b/extensions-core/kubernetes-extensions/src/test/java/org/apache/druid/k8s/discovery/K8sDruidNodeDiscoveryProviderTest.java index 4a9c5d041642..12a3a6118da8 100644 --- a/extensions-core/kubernetes-extensions/src/test/java/org/apache/druid/k8s/discovery/K8sDruidNodeDiscoveryProviderTest.java +++ b/extensions-core/kubernetes-extensions/src/test/java/org/apache/druid/k8s/discovery/K8sDruidNodeDiscoveryProviderTest.java @@ -240,7 +240,7 @@ public void testNodeRoleWatcherLoopOnNullItems() throws Exception ) ) ); - List> nullList = new ArrayList>(); + List> nullList = new ArrayList<>(); nullList.add(null); EasyMock.expect(mockK8sApiClient.watchPods( podInfo.getPodNamespace(), labelSelector, "v1", NodeRole.ROUTER)).andReturn( diff --git a/extensions-core/lookups-cached-global/pom.xml b/extensions-core/lookups-cached-global/pom.xml index 6fed0c9cb17b..287c6a0d89cf 100644 --- a/extensions-core/lookups-cached-global/pom.xml +++ b/extensions-core/lookups-cached-global/pom.xml @@ -84,11 +84,6 @@ jackson-core provided - - com.google.inject.extensions - guice-multibindings - provided - org.jdbi jdbi diff --git a/extensions-core/lookups-cached-global/src/main/java/org/apache/druid/data/input/MapPopulator.java b/extensions-core/lookups-cached-global/src/main/java/org/apache/druid/data/input/MapPopulator.java index c5e8c6e4bfc8..a0b591f21227 100644 --- a/extensions-core/lookups-cached-global/src/main/java/org/apache/druid/data/input/MapPopulator.java +++ b/extensions-core/lookups-cached-global/src/main/java/org/apache/druid/data/input/MapPopulator.java @@ -124,7 +124,7 @@ public PopulateResult populateAndWarnAtByteLimit( ) throws IOException { return source.asCharSource(StandardCharsets.UTF_8).readLines( - new LineProcessor() + new LineProcessor<>() { private int lines = 0; private int entries = 0; diff --git a/extensions-core/lookups-cached-global/src/main/java/org/apache/druid/query/lookup/namespace/UriExtractionNamespace.java b/extensions-core/lookups-cached-global/src/main/java/org/apache/druid/query/lookup/namespace/UriExtractionNamespace.java index e1c103c0ca23..75981f0b63cc 100644 --- a/extensions-core/lookups-cached-global/src/main/java/org/apache/druid/query/lookup/namespace/UriExtractionNamespace.java +++ b/extensions-core/lookups-cached-global/src/main/java/org/apache/druid/query/lookup/namespace/UriExtractionNamespace.java @@ -644,7 +644,7 @@ public ObjectMapperFlatDataParser( final JsonFactory jsonFactory = jsonMapper.getFactory().copy(); jsonFactory.configure(JsonFactory.Feature.CANONICALIZE_FIELD_NAMES, false); - parser = new Parser() + parser = new Parser<>() { @Override public Map parseToMap(String input) diff --git a/extensions-core/lookups-cached-global/src/main/java/org/apache/druid/server/lookup/namespace/NamespaceExtractionModule.java b/extensions-core/lookups-cached-global/src/main/java/org/apache/druid/server/lookup/namespace/NamespaceExtractionModule.java index 3aa087a4a1bb..2b25a877a43b 100644 --- a/extensions-core/lookups-cached-global/src/main/java/org/apache/druid/server/lookup/namespace/NamespaceExtractionModule.java +++ b/extensions-core/lookups-cached-global/src/main/java/org/apache/druid/server/lookup/namespace/NamespaceExtractionModule.java @@ -66,12 +66,8 @@ public static MapBinder, CacheGenerator> { return MapBinder.newMapBinder( binder, - new TypeLiteral>() - { - }, - new TypeLiteral>() - { - } + new TypeLiteral<>() {}, + new TypeLiteral<>() {} ); } diff --git a/extensions-core/lookups-cached-global/src/test/java/org/apache/druid/query/lookup/LookupUtilsTest.java b/extensions-core/lookups-cached-global/src/test/java/org/apache/druid/query/lookup/LookupUtilsTest.java index 7ef6ca501e50..503c97145c5e 100644 --- a/extensions-core/lookups-cached-global/src/test/java/org/apache/druid/query/lookup/LookupUtilsTest.java +++ b/extensions-core/lookups-cached-global/src/test/java/org/apache/druid/query/lookup/LookupUtilsTest.java @@ -42,14 +42,10 @@ public class LookupUtilsTest { private static final TypeReference> LOOKUPS_ALL_GENERIC_REFERENCE = - new TypeReference>() - { - }; + new TypeReference<>() {}; private static final TypeReference> LOOKUPS_ALL_REFERENCE = - new TypeReference>() - { - }; + new TypeReference<>() {}; private static final String LOOKUP_VALID_INNER = " \"lookup_uri_good\": {\n" + " \"version\": \"2021-12-03T01:04:05.151Z\",\n" diff --git a/extensions-core/lookups-cached-global/src/test/java/org/apache/druid/query/lookup/namespace/JSONFlatDataParserTest.java b/extensions-core/lookups-cached-global/src/test/java/org/apache/druid/query/lookup/namespace/JSONFlatDataParserTest.java index e44641924dc2..bb0c91e49533 100644 --- a/extensions-core/lookups-cached-global/src/test/java/org/apache/druid/query/lookup/namespace/JSONFlatDataParserTest.java +++ b/extensions-core/lookups-cached-global/src/test/java/org/apache/druid/query/lookup/namespace/JSONFlatDataParserTest.java @@ -70,7 +70,7 @@ public void setUp() throws Exception sink.writeLines( Iterables.transform( MAPPINGS, - new Function, CharSequence>() + new Function<>() { @Override public CharSequence apply(Map input) diff --git a/extensions-core/lookups-cached-global/src/test/java/org/apache/druid/server/lookup/namespace/UriCacheGeneratorTest.java b/extensions-core/lookups-cached-global/src/test/java/org/apache/druid/server/lookup/namespace/UriCacheGeneratorTest.java index 58ef2dc15de3..3caad9cec343 100644 --- a/extensions-core/lookups-cached-global/src/test/java/org/apache/druid/server/lookup/namespace/UriCacheGeneratorTest.java +++ b/extensions-core/lookups-cached-global/src/test/java/org/apache/druid/server/lookup/namespace/UriCacheGeneratorTest.java @@ -182,7 +182,7 @@ public void close() throws IOException new NamespaceExtractionConfig() ) ); - return () -> new Iterator() + return () -> new Iterator<>() { Iterator compressionIt = compressionParams.iterator(); Iterator> cacheManagerCreatorsIt = diff --git a/extensions-core/lookups-cached-global/src/test/java/org/apache/druid/server/lookup/namespace/cache/CacheSchedulerTest.java b/extensions-core/lookups-cached-global/src/test/java/org/apache/druid/server/lookup/namespace/cache/CacheSchedulerTest.java index fb4d6070e3e8..b6deed7be5ec 100644 --- a/extensions-core/lookups-cached-global/src/test/java/org/apache/druid/server/lookup/namespace/cache/CacheSchedulerTest.java +++ b/extensions-core/lookups-cached-global/src/test/java/org/apache/druid/server/lookup/namespace/cache/CacheSchedulerTest.java @@ -74,7 +74,7 @@ public class CacheSchedulerTest { public static final Function CREATE_ON_HEAP_CACHE_MANAGER = - new Function() + new Function<>() { @Nullable @Override @@ -88,7 +88,7 @@ public NamespaceExtractionCacheManager apply(@Nullable Lifecycle lifecycle) } }; public static final Function CREATE_OFF_HEAP_CACHE_MANAGER = - new Function() + new Function<>() { @Nullable @Override diff --git a/extensions-core/lookups-cached-single/src/main/java/org/apache/druid/server/lookup/cache/loading/OffHeapLoadingCache.java b/extensions-core/lookups-cached-single/src/main/java/org/apache/druid/server/lookup/cache/loading/OffHeapLoadingCache.java index 2d26ed7fdf82..2688dcaa0ef7 100644 --- a/extensions-core/lookups-cached-single/src/main/java/org/apache/druid/server/lookup/cache/loading/OffHeapLoadingCache.java +++ b/extensions-core/lookups-cached-single/src/main/java/org/apache/druid/server/lookup/cache/loading/OffHeapLoadingCache.java @@ -91,7 +91,7 @@ public OffHeapLoadingCache( .expireAfterAccess(this.expireAfterAccess, TimeUnit.MILLISECONDS) .expireMaxSize(this.maxEntriesSize) .make(); - cache.modificationListenerAdd(new Bind.MapListener() + cache.modificationListenerAdd(new Bind.MapListener<>() { @Override public void update(K key, V oldVal, V newVal) diff --git a/extensions-core/lookups-cached-single/src/main/java/org/apache/druid/server/lookup/cache/polling/OffHeapPollingCache.java b/extensions-core/lookups-cached-single/src/main/java/org/apache/druid/server/lookup/cache/polling/OffHeapPollingCache.java index b042e1ea1df9..a7cabf8d64fc 100644 --- a/extensions-core/lookups-cached-single/src/main/java/org/apache/druid/server/lookup/cache/polling/OffHeapPollingCache.java +++ b/extensions-core/lookups-cached-single/src/main/java/org/apache/druid/server/lookup/cache/polling/OffHeapPollingCache.java @@ -63,12 +63,12 @@ public OffHeapPollingCache(final Iterable> entries) final Set setOfValues = setOfValuesBuilder.build(); reverseCache.putAll(Maps.asMap( setOfValues, - new Function>() + new Function<>() { @Override public List apply(final V input) { - return Lists.newArrayList(Maps.filterKeys(mapCache, new Predicate() + return Lists.newArrayList(Maps.filterKeys(mapCache, new Predicate<>() { @Override public boolean apply(K key) diff --git a/extensions-core/lookups-cached-single/src/main/java/org/apache/druid/server/lookup/jdbc/QueryKeys.java b/extensions-core/lookups-cached-single/src/main/java/org/apache/druid/server/lookup/jdbc/QueryKeys.java index ca90fa668569..308191002044 100644 --- a/extensions-core/lookups-cached-single/src/main/java/org/apache/druid/server/lookup/jdbc/QueryKeys.java +++ b/extensions-core/lookups-cached-single/src/main/java/org/apache/druid/server/lookup/jdbc/QueryKeys.java @@ -55,7 +55,7 @@ public boolean accepts(Class type) @Override public ContainerBuilder> newContainerBuilderFor(Class type) { - return new ContainerBuilder>() + return new ContainerBuilder<>() { final ImmutableSet.Builder builder = new ImmutableSet.Builder<>(); diff --git a/extensions-core/lookups-cached-single/src/test/java/org/apache/druid/server/lookup/LoadingLookupFactoryTest.java b/extensions-core/lookups-cached-single/src/test/java/org/apache/druid/server/lookup/LoadingLookupFactoryTest.java index cf67e6e32eaf..4f98411e86d4 100644 --- a/extensions-core/lookups-cached-single/src/test/java/org/apache/druid/server/lookup/LoadingLookupFactoryTest.java +++ b/extensions-core/lookups-cached-single/src/test/java/org/apache/druid/server/lookup/LoadingLookupFactoryTest.java @@ -109,14 +109,14 @@ public void testSerDeser() throws IOException ObjectMapper mapper = TestHelper.makeJsonMapper(); LoadingLookupFactory loadingLookupFactory = new LoadingLookupFactory( new MockDataFetcher(), - new OnHeapLoadingCache( + new OnHeapLoadingCache<>( 0, 100, 100L, 0L, 0L ), - new OffHeapLoadingCache>( + new OffHeapLoadingCache<>( 100, 100L, 0L, diff --git a/extensions-core/multi-stage-query/pom.xml b/extensions-core/multi-stage-query/pom.xml index e2a7252908df..f80c4edaef83 100644 --- a/extensions-core/multi-stage-query/pom.xml +++ b/extensions-core/multi-stage-query/pom.xml @@ -71,11 +71,6 @@ guice provided - - com.google.inject.extensions - guice-multibindings - provided - com.google.guava guava diff --git a/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/dart/controller/sql/DartQueryMaker.java b/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/dart/controller/sql/DartQueryMaker.java index 66686f7640f9..c35ca39a9f04 100644 --- a/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/dart/controller/sql/DartQueryMaker.java +++ b/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/dart/controller/sql/DartQueryMaker.java @@ -237,7 +237,7 @@ private Sequence runWithReport(final ControllerHolder controllerHolder // Return a sequence that reads one row (the report) from reportFuture. return new BaseSequence<>( - new BaseSequence.IteratorMaker>() + new BaseSequence.IteratorMaker<>() { @Override public Iterator make() diff --git a/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/exec/ExceptionWrappingWorkerClient.java b/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/exec/ExceptionWrappingWorkerClient.java index 3373bbd883ee..daad4021368c 100644 --- a/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/exec/ExceptionWrappingWorkerClient.java +++ b/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/exec/ExceptionWrappingWorkerClient.java @@ -147,7 +147,7 @@ private static ListenableFuture wrap( Futures.addCallback( clientFuture, - new FutureCallback() + new FutureCallback<>() { @Override public void onSuccess(@Nullable T result) diff --git a/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/exec/ProcessingBuffersSet.java b/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/exec/ProcessingBuffersSet.java index 7f81a9c4a9c1..95f0335b71f5 100644 --- a/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/exec/ProcessingBuffersSet.java +++ b/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/exec/ProcessingBuffersSet.java @@ -74,7 +74,7 @@ public ResourceHolder acquireForStage(final StageDefinition s throw DruidException.defensive("Processing buffers not available"); } - return new ResourceHolder() + return new ResourceHolder<>() { @Override public ProcessingBuffers get() diff --git a/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/exec/RunWorkOrder.java b/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/exec/RunWorkOrder.java index 8b6216861a13..024036cdbff5 100644 --- a/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/exec/RunWorkOrder.java +++ b/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/exec/RunWorkOrder.java @@ -541,7 +541,7 @@ private void setUpCompletionCallbacks() stageOutputChannelsFuture ) ), - new FutureCallback>() + new FutureCallback<>() { @Override public void onSuccess(final List workerResultAndOutputChannelsResolved) @@ -1086,7 +1086,7 @@ private ResultAndChannels gatherResultKeyStatistics(final OutputChannels chan Futures.addCallback( clusterByStatisticsCollectorFuture, - new FutureCallback() + new FutureCallback<>() { @Override public void onSuccess(final ClusterByStatisticsCollector result) diff --git a/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/exec/WorkerImpl.java b/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/exec/WorkerImpl.java index 6c9f1d899413..44654810ac61 100644 --- a/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/exec/WorkerImpl.java +++ b/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/exec/WorkerImpl.java @@ -867,7 +867,7 @@ private ListeningExecutorService makeProcessingPool() @Override public Callable decorateCallable(Callable callable) { - return new PrioritizedCallable() + return new PrioritizedCallable<>() { @Override public int getPriority() diff --git a/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/exec/WorkerSketchFetcher.java b/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/exec/WorkerSketchFetcher.java index b0da5b83a46b..256e76307c81 100644 --- a/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/exec/WorkerSketchFetcher.java +++ b/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/exec/WorkerSketchFetcher.java @@ -156,7 +156,7 @@ private void fetchStatsFromWorker( SettableFuture kernelActionFuture = SettableFuture.create(); - Futures.addCallback(fetchFuture, new FutureCallback() + Futures.addCallback(fetchFuture, new FutureCallback<>() { @Override public void onSuccess(@Nullable ClusterByStatisticsSnapshot result) diff --git a/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/indexing/MSQCompactionRunner.java b/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/indexing/MSQCompactionRunner.java index d05ab12ea3fe..2d8a510fd98b 100644 --- a/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/indexing/MSQCompactionRunner.java +++ b/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/indexing/MSQCompactionRunner.java @@ -348,7 +348,10 @@ private static Integer getRowsPerSegment(CompactionTask compactionTask) private static RowSignature getRowSignature(DataSchema dataSchema) { RowSignature.Builder rowSignatureBuilder = RowSignature.builder(); - rowSignatureBuilder.add(dataSchema.getTimestampSpec().getTimestampColumn(), ColumnType.LONG); + if (dataSchema.getDimensionsSpec().isForceSegmentSortByTime() == true) { + // If sort not forced by time, __time appears as part of dimensions in DimensionsSpec + rowSignatureBuilder.add(dataSchema.getTimestampSpec().getTimestampColumn(), ColumnType.LONG); + } if (!isQueryGranularityEmptyOrNone(dataSchema)) { // A virtual column for query granularity would have been added. Add corresponding column type. rowSignatureBuilder.add(TIME_VIRTUAL_COLUMN, ColumnType.LONG); @@ -398,25 +401,31 @@ private static List getAggregateDimensions( private static ColumnMappings getColumnMappings(DataSchema dataSchema) { - List columnMappings = dataSchema.getDimensionsSpec() - .getDimensions() - .stream() - .map(dim -> new ColumnMapping( - dim.getName(), dim.getName())) - .collect(Collectors.toList()); + List columnMappings = new ArrayList<>(); + // For scan queries, a virtual column is created from __time if a custom query granularity is provided. For + // group-by queries, as insert needs __time, it will always be one of the dimensions. Since dimensions in groupby + // aren't allowed to have time column as the output name, we map time dimension to TIME_VIRTUAL_COLUMN in + // dimensions, and map it back to the time column here. + String timeColumn = (isGroupBy(dataSchema) || !isQueryGranularityEmptyOrNone(dataSchema)) + ? TIME_VIRTUAL_COLUMN + : ColumnHolder.TIME_COLUMN_NAME; + ColumnMapping timeColumnMapping = new ColumnMapping(timeColumn, ColumnHolder.TIME_COLUMN_NAME); + if (dataSchema.getDimensionsSpec().isForceSegmentSortByTime()) { + // When not sorted by time, the __time column is missing from dimensionsSpec + columnMappings.add(timeColumnMapping); + } + columnMappings.addAll( + dataSchema.getDimensionsSpec() + .getDimensions() + .stream() + .map(dim -> dim.getName().equals(ColumnHolder.TIME_COLUMN_NAME) + ? timeColumnMapping + : new ColumnMapping(dim.getName(), dim.getName())) + .collect(Collectors.toList()) + ); columnMappings.addAll(Arrays.stream(dataSchema.getAggregators()) .map(agg -> new ColumnMapping(agg.getName(), agg.getName())) - .collect( - Collectors.toList())); - if (isGroupBy(dataSchema) || !isQueryGranularityEmptyOrNone(dataSchema)) { - // For scan queries, a virtual column is created from __time if a custom query granularity is provided. For - // group-by queries, as insert needs __time, it will always be one of the dimensions. Since dimensions in groupby - // aren't allowed to have time column as the output name, we map time dimension to TIME_VIRTUAL_COLUMN in - // dimensions, and map it back to the time column here. - columnMappings.add(new ColumnMapping(TIME_VIRTUAL_COLUMN, ColumnHolder.TIME_COLUMN_NAME)); - } else { - columnMappings.add(new ColumnMapping(ColumnHolder.TIME_COLUMN_NAME, ColumnHolder.TIME_COLUMN_NAME)); - } + .collect(Collectors.toList())); return new ColumnMappings(columnMappings); } @@ -431,6 +440,19 @@ private static List getOrderBySpec(PartitionsSpec partitionSp return Collections.emptyList(); } + private static Map buildQueryContext( + Map taskContext, + DataSchema dataSchema + ) + { + if (dataSchema.getDimensionsSpec().isForceSegmentSortByTime()) { + return taskContext; + } + Map queryContext = new HashMap<>(taskContext); + queryContext.put(MultiStageQueryContext.CTX_FORCE_TIME_SORT, false); + return queryContext; + } + private static Query buildScanQuery( CompactionTask compactionTask, Interval interval, @@ -447,7 +469,7 @@ private static Query buildScanQuery( .columnTypes(rowSignature.getColumnTypes()) .intervals(new MultipleIntervalSegmentSpec(Collections.singletonList(interval))) .filters(dataSchema.getTransformSpec().getFilter()) - .context(compactionTask.getContext()); + .context(buildQueryContext(compactionTask.getContext(), dataSchema)); if (compactionTask.getTuningConfig() != null && compactionTask.getTuningConfig().getPartitionsSpec() != null) { List orderByColumnSpecs = getOrderBySpec(compactionTask.getTuningConfig().getPartitionsSpec()); @@ -599,7 +621,7 @@ private Query buildGroupByQuery( .setDimensions(getAggregateDimensions(dataSchema, inputColToVirtualCol)) .setAggregatorSpecs(Arrays.asList(dataSchema.getAggregators())) .setPostAggregatorSpecs(postAggregators) - .setContext(compactionTask.getContext()) + .setContext(buildQueryContext(compactionTask.getContext(), dataSchema)) .setInterval(interval); if (compactionTask.getTuningConfig() != null && compactionTask.getTuningConfig().getPartitionsSpec() != null) { diff --git a/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/indexing/error/BroadcastTablesTooLargeFault.java b/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/indexing/error/BroadcastTablesTooLargeFault.java index 6912fefa6b65..31337cbcfe69 100644 --- a/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/indexing/error/BroadcastTablesTooLargeFault.java +++ b/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/indexing/error/BroadcastTablesTooLargeFault.java @@ -24,7 +24,7 @@ import com.fasterxml.jackson.annotation.JsonProperty; import com.fasterxml.jackson.annotation.JsonTypeName; import org.apache.druid.java.util.common.StringUtils; -import org.apache.druid.sql.calcite.planner.JoinAlgorithm; +import org.apache.druid.query.JoinAlgorithm; import org.apache.druid.sql.calcite.planner.PlannerContext; import javax.annotation.Nullable; diff --git a/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/indexing/processor/SegmentGeneratorFrameProcessorFactory.java b/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/indexing/processor/SegmentGeneratorFrameProcessorFactory.java index 1796df89bf71..3fa5654c2a55 100644 --- a/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/indexing/processor/SegmentGeneratorFrameProcessorFactory.java +++ b/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/indexing/processor/SegmentGeneratorFrameProcessorFactory.java @@ -151,7 +151,7 @@ public ProcessorsAndChannels> makeProcessors( final Sequence> inputSequence = Sequences.simple(Iterables.transform( inputSliceReader.attach(0, slice, counters, warningPublisher), - new Function>() + new Function<>() { int i = 0; @@ -237,7 +237,7 @@ public boolean usesProcessingBuffers() @Override public TypeReference> getResultTypeReference() { - return new TypeReference>() {}; + return new TypeReference<>() {}; } @Nullable diff --git a/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/input/external/ExternalSegment.java b/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/input/external/ExternalSegment.java index 4b69a930b1a6..3aae435b8d99 100644 --- a/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/input/external/ExternalSegment.java +++ b/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/input/external/ExternalSegment.java @@ -76,7 +76,7 @@ public CloseableIterator make() { try { CloseableIterator baseIterator = reader.read(inputStats); - return new CloseableIterator() + return new CloseableIterator<>() { private InputRow next = null; diff --git a/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/querykit/BaseFrameProcessorFactory.java b/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/querykit/BaseFrameProcessorFactory.java index 33f702aca0cc..4540af086cd5 100644 --- a/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/querykit/BaseFrameProcessorFactory.java +++ b/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/querykit/BaseFrameProcessorFactory.java @@ -37,7 +37,7 @@ public abstract class BaseFrameProcessorFactory implements FrameProcessorFactory @Override public TypeReference getResultTypeReference() { - return new TypeReference() {}; + return new TypeReference<>() {}; } @Override diff --git a/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/querykit/BroadcastJoinSegmentMapFnProcessor.java b/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/querykit/BroadcastJoinSegmentMapFnProcessor.java index cbb79c45702b..b47a450aa000 100644 --- a/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/querykit/BroadcastJoinSegmentMapFnProcessor.java +++ b/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/querykit/BroadcastJoinSegmentMapFnProcessor.java @@ -38,11 +38,11 @@ import org.apache.druid.msq.input.ReadableInput; import org.apache.druid.query.DataSource; import org.apache.druid.query.InlineDataSource; +import org.apache.druid.query.JoinAlgorithm; import org.apache.druid.query.Query; import org.apache.druid.segment.ColumnValueSelector; import org.apache.druid.segment.Cursor; import org.apache.druid.segment.SegmentReference; -import org.apache.druid.sql.calcite.planner.JoinAlgorithm; import org.apache.druid.sql.calcite.planner.PlannerContext; import java.io.IOException; diff --git a/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/querykit/DataSourcePlan.java b/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/querykit/DataSourcePlan.java index 6b8fbcb6f352..d20adc1e0062 100644 --- a/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/querykit/DataSourcePlan.java +++ b/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/querykit/DataSourcePlan.java @@ -47,6 +47,7 @@ import org.apache.druid.query.DataSource; import org.apache.druid.query.FilteredDataSource; import org.apache.druid.query.InlineDataSource; +import org.apache.druid.query.JoinAlgorithm; import org.apache.druid.query.JoinDataSource; import org.apache.druid.query.LookupDataSource; import org.apache.druid.query.QueryContext; @@ -65,8 +66,6 @@ import org.apache.druid.segment.join.JoinConditionAnalysis; import org.apache.druid.sql.calcite.external.ExternalDataSource; import org.apache.druid.sql.calcite.parser.DruidSqlInsert; -import org.apache.druid.sql.calcite.planner.JoinAlgorithm; -import org.apache.druid.sql.calcite.planner.PlannerContext; import org.joda.time.Interval; import javax.annotation.Nullable; @@ -212,10 +211,11 @@ public static DataSourcePlan forDataSource( broadcast ); } else if (dataSource instanceof JoinDataSource) { - final JoinAlgorithm preferredJoinAlgorithm = PlannerContext.getJoinAlgorithm(queryContext); + JoinDataSource joinDataSource = (JoinDataSource) dataSource; + final JoinAlgorithm preferredJoinAlgorithm = joinDataSource.getJoinAlgorithm(); final JoinAlgorithm deducedJoinAlgorithm = deduceJoinAlgorithm( preferredJoinAlgorithm, - ((JoinDataSource) dataSource) + joinDataSource ); switch (deducedJoinAlgorithm) { @@ -223,7 +223,7 @@ public static DataSourcePlan forDataSource( return forBroadcastHashJoin( queryKitSpec, queryContext, - (JoinDataSource) dataSource, + joinDataSource, querySegmentSpec, filter, filterFields, @@ -234,7 +234,7 @@ public static DataSourcePlan forDataSource( case SORT_MERGE: return forSortMergeJoin( queryKitSpec, - (JoinDataSource) dataSource, + joinDataSource, querySegmentSpec, minStageNumber, broadcast @@ -615,7 +615,8 @@ private static DataSourcePlan forBroadcastHashJoin( clause.getJoinType(), // First JoinDataSource (i == 0) involves the base table, so we need to propagate the base table filter. i == 0 ? analysis.getJoinBaseTableFilter().orElse(null) : null, - dataSource.getJoinableFactoryWrapper() + dataSource.getJoinableFactoryWrapper(), + clause.getJoinAlgorithm() ); inputSpecs.addAll(clausePlan.getInputSpecs()); clausePlan.getBroadcastInputs().intStream().forEach(n -> broadcastInputs.add(n + shift)); diff --git a/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/querykit/results/ExportResultsFrameProcessorFactory.java b/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/querykit/results/ExportResultsFrameProcessorFactory.java index d4ba2120c9c3..468f18214236 100644 --- a/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/querykit/results/ExportResultsFrameProcessorFactory.java +++ b/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/querykit/results/ExportResultsFrameProcessorFactory.java @@ -184,7 +184,7 @@ public ProcessorsAndChannels makeProcessors( @Override public TypeReference getResultTypeReference() { - return new TypeReference() {}; + return new TypeReference<>() {}; } @Override diff --git a/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/rpc/BaseWorkerClientImpl.java b/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/rpc/BaseWorkerClientImpl.java index d3428a6c3428..8e4f9678d056 100644 --- a/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/rpc/BaseWorkerClientImpl.java +++ b/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/rpc/BaseWorkerClientImpl.java @@ -190,7 +190,7 @@ public ListenableFuture getCounters(String workerId) new RequestBuilder(HttpMethod.GET, "/counters").header(HttpHeaders.ACCEPT, contentType), new BytesFullResponseHandler() ), - holder -> deserialize(holder, new TypeReference() {}) + holder -> deserialize(holder, new TypeReference<>() {}) ); } @@ -216,7 +216,7 @@ public ListenableFuture fetchChannelData( Futures.addCallback( clientFuture, - new FutureCallback() + new FutureCallback<>() { @Override public void onSuccess(FrameFilePartialFetch partialFetch) diff --git a/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/rpc/WorkerResource.java b/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/rpc/WorkerResource.java index fd6207662af8..0cc784cf36dd 100644 --- a/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/rpc/WorkerResource.java +++ b/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/rpc/WorkerResource.java @@ -144,7 +144,7 @@ public void onStartAsync(AsyncEvent event) Futures.addCallback( dataFuture, - new FutureCallback() + new FutureCallback<>() { @Override public void onSuccess(final InputStream inputStream) diff --git a/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/shuffle/input/WorkerInputChannelFactory.java b/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/shuffle/input/WorkerInputChannelFactory.java index 4dde34a67f59..84bf1cccb9d4 100644 --- a/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/shuffle/input/WorkerInputChannelFactory.java +++ b/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/shuffle/input/WorkerInputChannelFactory.java @@ -74,7 +74,7 @@ private void fetch( Futures.addCallback( fetchFuture, - new FutureCallback() + new FutureCallback<>() { @Override public void onSuccess(final Boolean lastFetch) diff --git a/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/sql/resources/SqlTaskResource.java b/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/sql/resources/SqlTaskResource.java index a07ef2c403fb..11e39bfa1267 100644 --- a/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/sql/resources/SqlTaskResource.java +++ b/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/sql/resources/SqlTaskResource.java @@ -34,6 +34,7 @@ import org.apache.druid.msq.guice.MultiStageQuery; import org.apache.druid.msq.sql.MSQTaskSqlEngine; import org.apache.druid.query.QueryException; +import org.apache.druid.query.http.SqlTaskStatus; import org.apache.druid.server.QueryResponse; import org.apache.druid.server.initialization.ServerConfig; import org.apache.druid.server.security.Access; @@ -47,7 +48,6 @@ import org.apache.druid.sql.http.ResultFormat; import org.apache.druid.sql.http.SqlQuery; import org.apache.druid.sql.http.SqlResource; -import org.apache.druid.sql.http.SqlTaskStatus; import javax.servlet.http.HttpServletRequest; import javax.ws.rs.Consumes; diff --git a/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/util/MSQFutureUtils.java b/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/util/MSQFutureUtils.java index 1635a7964198..15f8d3143483 100644 --- a/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/util/MSQFutureUtils.java +++ b/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/util/MSQFutureUtils.java @@ -47,7 +47,7 @@ public static ListenableFuture> allAsList( if (cancelOnErrorOrInterrupt) { Futures.addCallback( retVal, - new FutureCallback>() + new FutureCallback<>() { @Override public void onSuccess(@Nullable List result) diff --git a/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/util/MultiStageQueryContext.java b/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/util/MultiStageQueryContext.java index 7112a101c04f..f3cb66fd27a4 100644 --- a/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/util/MultiStageQueryContext.java +++ b/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/util/MultiStageQueryContext.java @@ -448,9 +448,7 @@ static List decodeList(final String keyName, @Nullable final String list try { // Not caching this ObjectMapper in a static, because we expect to use it infrequently (once per INSERT // query that uses this feature) and there is no need to keep it around longer than that. - return new ObjectMapper().readValue(listString, new TypeReference>() - { - }); + return new ObjectMapper().readValue(listString, new TypeReference<>() {}); } catch (JsonProcessingException e) { throw QueryContexts.badValueException(keyName, "CSV or JSON array", listString); diff --git a/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/util/SqlStatementResourceHelper.java b/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/util/SqlStatementResourceHelper.java index 0820342ba727..f70dd81caee0 100644 --- a/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/util/SqlStatementResourceHelper.java +++ b/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/util/SqlStatementResourceHelper.java @@ -321,7 +321,7 @@ public static Sequence getResultSequence( .map(mapping -> columnSelectorFactory.makeColumnValueSelector(mapping.getQueryColumn())) .collect(Collectors.toList()); - final Iterable retVal = () -> new Iterator() + final Iterable retVal = () -> new Iterator<>() { @Override public boolean hasNext() diff --git a/extensions-core/multi-stage-query/src/test/java/org/apache/druid/msq/dart/controller/http/DartSqlResourceTest.java b/extensions-core/multi-stage-query/src/test/java/org/apache/druid/msq/dart/controller/http/DartSqlResourceTest.java index 51e17235203d..981f96fbe2a8 100644 --- a/extensions-core/multi-stage-query/src/test/java/org/apache/druid/msq/dart/controller/http/DartSqlResourceTest.java +++ b/extensions-core/multi-stage-query/src/test/java/org/apache/druid/msq/dart/controller/http/DartSqlResourceTest.java @@ -571,7 +571,7 @@ public void test_doPost_regularUser_fullReport() throws Exception final List> reportMaps = objectMapper.readValue( asyncResponse.baos.toByteArray(), - new TypeReference>>() {} + new TypeReference<>() {} ); Assertions.assertEquals(1, reportMaps.size()); @@ -610,7 +610,7 @@ public void test_doPost_regularUser_runtimeError_fullReport() throws Exception final List> reportMaps = objectMapper.readValue( asyncResponse.baos.toByteArray(), - new TypeReference>>() {} + new TypeReference<>() {} ); Assertions.assertEquals(1, reportMaps.size()); diff --git a/extensions-core/multi-stage-query/src/test/java/org/apache/druid/msq/exec/MSQSelectTest.java b/extensions-core/multi-stage-query/src/test/java/org/apache/druid/msq/exec/MSQSelectTest.java index 09dcbfcda144..6b904e5f1495 100644 --- a/extensions-core/multi-stage-query/src/test/java/org/apache/druid/msq/exec/MSQSelectTest.java +++ b/extensions-core/multi-stage-query/src/test/java/org/apache/druid/msq/exec/MSQSelectTest.java @@ -50,6 +50,7 @@ import org.apache.druid.msq.test.MSQTestBase; import org.apache.druid.msq.util.MultiStageQueryContext; import org.apache.druid.query.InlineDataSource; +import org.apache.druid.query.JoinAlgorithm; import org.apache.druid.query.LookupDataSource; import org.apache.druid.query.OrderBy; import org.apache.druid.query.Query; @@ -81,7 +82,6 @@ import org.apache.druid.sql.calcite.filtration.Filtration; import org.apache.druid.sql.calcite.planner.ColumnMapping; import org.apache.druid.sql.calcite.planner.ColumnMappings; -import org.apache.druid.sql.calcite.planner.JoinAlgorithm; import org.apache.druid.sql.calcite.planner.PlannerContext; import org.apache.druid.sql.calcite.util.CalciteTests; import org.hamcrest.CoreMatchers; @@ -1043,7 +1043,9 @@ private void testJoin(String contextName, Map context, final Joi DruidExpression.ofColumn(ColumnType.FLOAT, "m1"), DruidExpression.ofColumn(ColumnType.FLOAT, "j0.m1") ), - JoinType.INNER + JoinType.INNER, + null, + joinAlgorithm ) ) .setInterval(querySegmentSpec(Filtration.eternity())) @@ -2523,7 +2525,9 @@ public void testJoinUsesDifferentAlgorithm(String contextName, Map DIMENSIONS = ImmutableList.of( STRING_DIMENSION, LONG_DIMENSION, - MV_STRING_DIMENSION + MV_STRING_DIMENSION, + NESTED_DIMENSION, + AUTO_DIMENSION ); private static final Map INTERVAL_DATASCHEMAS = ImmutableMap.of( COMPACTION_INTERVAL, @@ -144,7 +149,6 @@ public void testMultipleDisjointCompactionIntervalsAreInvalid() null, Collections.emptyMap(), null, - null, null ); CompactionConfigValidationResult validationResult = MSQ_COMPACTION_RUNNER.validateCompactionTask( @@ -166,7 +170,6 @@ public void testHashedPartitionsSpecIsInvalid() null, Collections.emptyMap(), null, - null, null ); Assert.assertFalse(MSQ_COMPACTION_RUNNER.validateCompactionTask(compactionTask, INTERVAL_DATASCHEMAS).isValid()); @@ -180,7 +183,6 @@ public void testStringDimensionInRangePartitionsSpecIsValid() null, Collections.emptyMap(), null, - null, null ); Assert.assertTrue(MSQ_COMPACTION_RUNNER.validateCompactionTask(compactionTask, INTERVAL_DATASCHEMAS).isValid()); @@ -195,7 +197,6 @@ public void testLongDimensionInRangePartitionsSpecIsInvalid() null, Collections.emptyMap(), null, - null, null ); @@ -218,7 +219,6 @@ public void testMultiValuedDimensionInRangePartitionsSpecIsInvalid() null, Collections.emptyMap(), null, - null, null ); @@ -240,7 +240,6 @@ public void testMaxTotalRowsIsInvalid() null, Collections.emptyMap(), null, - null, null ); Assert.assertFalse(MSQ_COMPACTION_RUNNER.validateCompactionTask(compactionTask, INTERVAL_DATASCHEMAS).isValid()); @@ -254,7 +253,6 @@ public void testDynamicPartitionsSpecIsValid() null, Collections.emptyMap(), null, - null, null ); Assert.assertTrue(MSQ_COMPACTION_RUNNER.validateCompactionTask(compactionTask, INTERVAL_DATASCHEMAS).isValid()); @@ -268,7 +266,6 @@ public void testQueryGranularityAllIsValid() null, Collections.emptyMap(), new ClientCompactionTaskGranularitySpec(null, Granularities.ALL, null), - null, null ); Assert.assertTrue(MSQ_COMPACTION_RUNNER.validateCompactionTask(compactionTask, INTERVAL_DATASCHEMAS).isValid()); @@ -282,7 +279,6 @@ public void testRollupFalseWithMetricsSpecIsInValid() null, Collections.emptyMap(), new ClientCompactionTaskGranularitySpec(null, null, false), - null, AGGREGATORS.toArray(new AggregatorFactory[0]) ); Assert.assertFalse(MSQ_COMPACTION_RUNNER.validateCompactionTask(compactionTask, INTERVAL_DATASCHEMAS).isValid()); @@ -296,7 +292,6 @@ public void testRollupTrueWithoutMetricsSpecIsInvalid() null, Collections.emptyMap(), new ClientCompactionTaskGranularitySpec(null, null, true), - null, null ); Assert.assertFalse(MSQ_COMPACTION_RUNNER.validateCompactionTask(compactionTask, INTERVAL_DATASCHEMAS).isValid()); @@ -313,7 +308,6 @@ public void testMSQEngineWithUnsupportedMetricsSpecIsInvalid() null, Collections.emptyMap(), new ClientCompactionTaskGranularitySpec(null, null, true), - null, new AggregatorFactory[]{new LongSumAggregatorFactory(outputColName, inputColName)} ); CompactionConfigValidationResult validationResult = MSQ_COMPACTION_RUNNER.validateCompactionTask( @@ -330,13 +324,13 @@ public void testMSQEngineWithUnsupportedMetricsSpecIsInvalid() @Test public void testRunCompactionTasksWithEmptyTaskListFails() throws Exception { - CompactionTask compactionTask = createCompactionTask(null, null, Collections.emptyMap(), null, null, null); + CompactionTask compactionTask = createCompactionTask(null, null, Collections.emptyMap(), null, null); TaskStatus taskStatus = MSQ_COMPACTION_RUNNER.runCompactionTasks(compactionTask, Collections.emptyMap(), null); Assert.assertTrue(taskStatus.isFailure()); } @Test - public void testMSQControllerTaskSpecWithScanIsValid() throws JsonProcessingException + public void testCompactionConfigWithoutMetricsSpecProducesCorrectSpec() throws JsonProcessingException { DimFilter dimFilter = new SelectorDimFilter("dim1", "foo", null); @@ -345,7 +339,6 @@ public void testMSQControllerTaskSpecWithScanIsValid() throws JsonProcessingExce dimFilter, Collections.emptyMap(), null, - null, null ); @@ -357,7 +350,7 @@ public void testMSQControllerTaskSpecWithScanIsValid() throws JsonProcessingExce .withGranularity( new UniformGranularitySpec( SEGMENT_GRANULARITY.getDefaultGranularity(), - null, + QUERY_GRANULARITY.getDefaultGranularity(), false, Collections.singletonList(COMPACTION_INTERVAL) ) @@ -375,37 +368,37 @@ public void testMSQControllerTaskSpecWithScanIsValid() throws JsonProcessingExce MSQSpec actualMSQSpec = msqControllerTask.getQuerySpec(); - Assert.assertEquals( - new MSQTuningConfig( - 1, - MultiStageQueryContext.DEFAULT_ROWS_IN_MEMORY, - MAX_ROWS_PER_SEGMENT, - null, - createIndexSpec() - ), - actualMSQSpec.getTuningConfig() - ); - Assert.assertEquals( - new DataSourceMSQDestination( - DATA_SOURCE, - SEGMENT_GRANULARITY.getDefaultGranularity(), - null, - Collections.singletonList(COMPACTION_INTERVAL), - DIMENSIONS.stream().collect(Collectors.toMap(DimensionSchema::getName, Function.identity())), - null - ), - actualMSQSpec.getDestination() - ); + Assert.assertEquals(getExpectedTuningConfig(), actualMSQSpec.getTuningConfig()); + Assert.assertEquals(getExpectedDestination(), actualMSQSpec.getDestination()); Assert.assertTrue(actualMSQSpec.getQuery() instanceof ScanQuery); ScanQuery scanQuery = (ScanQuery) actualMSQSpec.getQuery(); + List expectedColumns = new ArrayList<>(); + List expectedColumnTypes = new ArrayList<>(); + // Add __time since this is a time-ordered query which doesn't have __time explicitly defined in dimensionsSpec + expectedColumns.add(ColumnHolder.TIME_COLUMN_NAME); + expectedColumnTypes.add(ColumnType.LONG); + + // Add TIME_VIRTUAL_COLUMN since a query granularity is specified + expectedColumns.add(MSQCompactionRunner.TIME_VIRTUAL_COLUMN); + expectedColumnTypes.add(ColumnType.LONG); + + expectedColumns.addAll(DIMENSIONS.stream().map(DimensionSchema::getName).collect(Collectors.toList())); + expectedColumnTypes.addAll(DIMENSIONS.stream().map(DimensionSchema::getColumnType).collect(Collectors.toList())); + + Assert.assertEquals(expectedColumns, scanQuery.getColumns()); + Assert.assertEquals(expectedColumnTypes, scanQuery.getColumnTypes()); + Assert.assertEquals(dimFilter, scanQuery.getFilter()); Assert.assertEquals( JSON_MAPPER.writeValueAsString(SEGMENT_GRANULARITY.toString()), msqControllerTask.getContext().get(DruidSqlInsert.SQL_INSERT_SEGMENT_GRANULARITY) ); - Assert.assertNull(msqControllerTask.getContext().get(DruidSqlInsert.SQL_INSERT_QUERY_GRANULARITY)); + Assert.assertEquals( + JSON_MAPPER.writeValueAsString(QUERY_GRANULARITY.toString()), + msqControllerTask.getContext().get(DruidSqlInsert.SQL_INSERT_QUERY_GRANULARITY) + ); Assert.assertEquals(WorkerAssignmentStrategy.MAX, actualMSQSpec.getAssignmentStrategy()); Assert.assertEquals( PARTITION_DIMENSIONS.stream().map(OrderBy::ascending).collect(Collectors.toList()), @@ -414,7 +407,60 @@ public void testMSQControllerTaskSpecWithScanIsValid() throws JsonProcessingExce } @Test - public void testMSQControllerTaskSpecWithAggregatorsIsValid() throws JsonProcessingException + public void testCompactionConfigWithSortOnNonTimeDimensionsProducesCorrectSpec() throws JsonProcessingException + { + List nonTimeSortedDimensions = ImmutableList.of( + STRING_DIMENSION, + new LongDimensionSchema(ColumnHolder.TIME_COLUMN_NAME), + LONG_DIMENSION + ); + CompactionTask taskCreatedWithTransformSpec = createCompactionTask( + new DynamicPartitionsSpec(TARGET_ROWS_PER_SEGMENT, null), + null, + Collections.emptyMap(), + null, + null + ); + + // Set forceSegmentSortByTime=false to enable non-time order + DimensionsSpec dimensionsSpec = DimensionsSpec.builder() + .setDimensions(nonTimeSortedDimensions) + .setForceSegmentSortByTime(false) + .build(); + DataSchema dataSchema = + DataSchema.builder() + .withDataSource(DATA_SOURCE) + .withTimestamp(new TimestampSpec(TIMESTAMP_COLUMN, null, null)) + .withDimensions(dimensionsSpec) + .withGranularity( + new UniformGranularitySpec( + SEGMENT_GRANULARITY.getDefaultGranularity(), + null, + false, + Collections.singletonList(COMPACTION_INTERVAL) + ) + ) + .build(); + + List msqControllerTasks = MSQ_COMPACTION_RUNNER.createMsqControllerTasks( + taskCreatedWithTransformSpec, + Collections.singletonMap(COMPACTION_INTERVAL, dataSchema) + ); + + MSQSpec actualMSQSpec = Iterables.getOnlyElement(msqControllerTasks).getQuerySpec(); + + Assert.assertTrue(actualMSQSpec.getQuery() instanceof ScanQuery); + ScanQuery scanQuery = (ScanQuery) actualMSQSpec.getQuery(); + + // Dimensions should already list __time and the order should remain intact + Assert.assertEquals( + nonTimeSortedDimensions.stream().map(DimensionSchema::getName).collect(Collectors.toList()), + scanQuery.getColumns() + ); + } + + @Test + public void testCompactionConfigWithMetricsSpecProducesCorrectSpec() throws JsonProcessingException { DimFilter dimFilter = new SelectorDimFilter("dim1", "foo", null); @@ -423,7 +469,6 @@ public void testMSQControllerTaskSpecWithAggregatorsIsValid() throws JsonProcess dimFilter, Collections.emptyMap(), null, - null, null ); @@ -444,7 +489,6 @@ public void testMSQControllerTaskSpecWithAggregatorsIsValid() throws JsonProcess multiValuedDimensions ); - List msqControllerTasks = MSQ_COMPACTION_RUNNER.createMsqControllerTasks( taskCreatedWithTransformSpec, Collections.singletonMap(COMPACTION_INTERVAL, dataSchema) @@ -454,27 +498,8 @@ public void testMSQControllerTaskSpecWithAggregatorsIsValid() throws JsonProcess MSQSpec actualMSQSpec = msqControllerTask.getQuerySpec(); - Assert.assertEquals( - new MSQTuningConfig( - 1, - MultiStageQueryContext.DEFAULT_ROWS_IN_MEMORY, - MAX_ROWS_PER_SEGMENT, - null, - createIndexSpec() - ), - actualMSQSpec.getTuningConfig() - ); - Assert.assertEquals( - new DataSourceMSQDestination( - DATA_SOURCE, - SEGMENT_GRANULARITY.getDefaultGranularity(), - null, - Collections.singletonList(COMPACTION_INTERVAL), - DIMENSIONS.stream().collect(Collectors.toMap(DimensionSchema::getName, Function.identity())), - null - ), - actualMSQSpec.getDestination() - ); + Assert.assertEquals(getExpectedTuningConfig(), actualMSQSpec.getTuningConfig()); + Assert.assertEquals(getExpectedDestination(), actualMSQSpec.getDestination()); Assert.assertTrue(actualMSQSpec.getQuery() instanceof GroupByQuery); GroupByQuery groupByQuery = (GroupByQuery) actualMSQSpec.getQuery(); @@ -490,30 +515,32 @@ public void testMSQControllerTaskSpecWithAggregatorsIsValid() throws JsonProcess ); Assert.assertEquals(WorkerAssignmentStrategy.MAX, actualMSQSpec.getAssignmentStrategy()); - - // Since only MV_STRING_DIMENSION is indicated to be MVD by the CombinedSchema, conversion to array should happen - // only for that column. - List expectedDimensionSpec = DIMENSIONS.stream() - .filter(dim -> !MV_STRING_DIMENSION.getName() - .equals(dim.getName())) - .map(dim -> new DefaultDimensionSpec( - dim.getName(), - dim.getName(), - dim.getColumnType() - )) - .collect( - Collectors.toList()); + List expectedDimensionSpec = new ArrayList<>(); expectedDimensionSpec.add( - new DefaultDimensionSpec(MSQCompactionRunner.TIME_VIRTUAL_COLUMN, - MSQCompactionRunner.TIME_VIRTUAL_COLUMN, - ColumnType.LONG) + new DefaultDimensionSpec( + MSQCompactionRunner.TIME_VIRTUAL_COLUMN, + MSQCompactionRunner.TIME_VIRTUAL_COLUMN, + ColumnType.LONG + ) ); String mvToArrayStringDim = MSQCompactionRunner.ARRAY_VIRTUAL_COLUMN_PREFIX + MV_STRING_DIMENSION.getName(); - expectedDimensionSpec.add(new DefaultDimensionSpec(mvToArrayStringDim, mvToArrayStringDim, ColumnType.STRING_ARRAY)); - MatcherAssert.assertThat( - expectedDimensionSpec, - Matchers.containsInAnyOrder(groupByQuery.getDimensions().toArray(new DimensionSpec[0])) - ); + // Since only MV_STRING_DIMENSION is indicated to be MVD by the CombinedSchema, conversion to array should happen + // only for that column. + expectedDimensionSpec.addAll(DIMENSIONS.stream() + .map(dim -> + MV_STRING_DIMENSION.getName().equals(dim.getName()) + ? new DefaultDimensionSpec( + mvToArrayStringDim, + mvToArrayStringDim, + ColumnType.STRING_ARRAY + ) + : new DefaultDimensionSpec( + dim.getName(), + dim.getName(), + dim.getColumnType() + )) + .collect(Collectors.toList())); + Assert.assertEquals(expectedDimensionSpec, groupByQuery.getDimensions()); } private CompactionTask createCompactionTask( @@ -521,7 +548,6 @@ private CompactionTask createCompactionTask( @Nullable DimFilter dimFilter, Map contextParams, @Nullable ClientCompactionTaskGranularitySpec granularitySpec, - @Nullable List dimensionSchemas, @Nullable AggregatorFactory[] metricsSpec ) { @@ -545,7 +571,7 @@ private CompactionTask createCompactionTask( )) .transformSpec(transformSpec) .granularitySpec(granularitySpec) - .dimensionsSpec(new DimensionsSpec(dimensionSchemas)) + .dimensionsSpec(new DimensionsSpec(null)) .metricsSpec(metricsSpec) .compactionRunner(MSQ_COMPACTION_RUNNER) .context(context); @@ -580,4 +606,27 @@ private static IndexSpec createIndexSpec() .withLongEncoding(CompressionFactory.LongEncodingStrategy.LONGS) .build(); } + + private static DataSourceMSQDestination getExpectedDestination() + { + return new DataSourceMSQDestination( + DATA_SOURCE, + SEGMENT_GRANULARITY.getDefaultGranularity(), + null, + Collections.singletonList(COMPACTION_INTERVAL), + DIMENSIONS.stream().collect(Collectors.toMap(DimensionSchema::getName, Function.identity())), + null + ); + } + + private static MSQTuningConfig getExpectedTuningConfig() + { + return new MSQTuningConfig( + 1, + MultiStageQueryContext.DEFAULT_ROWS_IN_MEMORY, + MAX_ROWS_PER_SEGMENT, + null, + createIndexSpec() + ); + } } diff --git a/extensions-core/multi-stage-query/src/test/java/org/apache/druid/msq/indexing/TaskReportQueryListenerTest.java b/extensions-core/multi-stage-query/src/test/java/org/apache/druid/msq/indexing/TaskReportQueryListenerTest.java index 11c33d215170..252419f7227a 100644 --- a/extensions-core/multi-stage-query/src/test/java/org/apache/druid/msq/indexing/TaskReportQueryListenerTest.java +++ b/extensions-core/multi-stage-query/src/test/java/org/apache/druid/msq/indexing/TaskReportQueryListenerTest.java @@ -114,7 +114,7 @@ public void test_taskReportDestination() throws IOException final TaskReport.ReportMap reportMap = JSON_MAPPER.readValue( baos.toByteArray(), - new TypeReference() {} + new TypeReference<>() {} ); Assert.assertEquals(ImmutableSet.of("multiStageQuery", TaskContextReport.REPORT_KEY), reportMap.keySet()); @@ -183,7 +183,7 @@ public void test_durableDestination() throws IOException final TaskReport.ReportMap reportMap = JSON_MAPPER.readValue( baos.toByteArray(), - new TypeReference() {} + new TypeReference<>() {} ); Assert.assertEquals(ImmutableSet.of("multiStageQuery", TaskContextReport.REPORT_KEY), reportMap.keySet()); diff --git a/extensions-core/multi-stage-query/src/test/java/org/apache/druid/msq/indexing/error/MSQFaultSerdeTest.java b/extensions-core/multi-stage-query/src/test/java/org/apache/druid/msq/indexing/error/MSQFaultSerdeTest.java index 55c6c48c1afe..3d26444a3d27 100644 --- a/extensions-core/multi-stage-query/src/test/java/org/apache/druid/msq/indexing/error/MSQFaultSerdeTest.java +++ b/extensions-core/multi-stage-query/src/test/java/org/apache/druid/msq/indexing/error/MSQFaultSerdeTest.java @@ -26,9 +26,9 @@ import org.apache.druid.java.util.common.Intervals; import org.apache.druid.java.util.common.granularity.Granularities; import org.apache.druid.msq.guice.MSQIndexingModule; +import org.apache.druid.query.JoinAlgorithm; import org.apache.druid.segment.TestHelper; import org.apache.druid.segment.column.ColumnType; -import org.apache.druid.sql.calcite.planner.JoinAlgorithm; import org.junit.Assert; import org.junit.Before; import org.junit.Test; diff --git a/extensions-core/multi-stage-query/src/test/java/org/apache/druid/msq/querykit/BroadcastJoinSegmentMapFnProcessorTest.java b/extensions-core/multi-stage-query/src/test/java/org/apache/druid/msq/querykit/BroadcastJoinSegmentMapFnProcessorTest.java index 4270fe8bdccc..2f3efa679cd8 100644 --- a/extensions-core/multi-stage-query/src/test/java/org/apache/druid/msq/querykit/BroadcastJoinSegmentMapFnProcessorTest.java +++ b/extensions-core/multi-stage-query/src/test/java/org/apache/druid/msq/querykit/BroadcastJoinSegmentMapFnProcessorTest.java @@ -42,6 +42,7 @@ import org.apache.druid.msq.indexing.error.MSQException; import org.apache.druid.query.DataSource; import org.apache.druid.query.InlineDataSource; +import org.apache.druid.query.JoinAlgorithm; import org.apache.druid.query.JoinDataSource; import org.apache.druid.query.Query; import org.apache.druid.query.QueryContext; @@ -50,7 +51,6 @@ import org.apache.druid.segment.TestIndex; import org.apache.druid.segment.join.JoinConditionAnalysis; import org.apache.druid.segment.join.JoinType; -import org.apache.druid.sql.calcite.planner.JoinAlgorithm; import org.apache.druid.sql.calcite.planner.PlannerContext; import org.apache.druid.testing.InitializedNullHandlingTest; import org.easymock.EasyMock; @@ -190,6 +190,7 @@ public void testBuildTableAndInlineData() throws IOException JoinConditionAnalysis.forExpression("x == \"j.x\"", "j.", ExprMacroTable.nil()), JoinType.INNER, null, + null, null ) ); diff --git a/extensions-core/multi-stage-query/src/test/java/org/apache/druid/msq/querykit/scan/ScanQueryFrameProcessorTest.java b/extensions-core/multi-stage-query/src/test/java/org/apache/druid/msq/querykit/scan/ScanQueryFrameProcessorTest.java index 96957da53213..21e3707f00fd 100644 --- a/extensions-core/multi-stage-query/src/test/java/org/apache/druid/msq/querykit/scan/ScanQueryFrameProcessorTest.java +++ b/extensions-core/multi-stage-query/src/test/java/org/apache/druid/msq/querykit/scan/ScanQueryFrameProcessorTest.java @@ -120,7 +120,7 @@ public void test_runWithSegments() throws Exception ) ), Function.identity(), - new ResourceHolder() + new ResourceHolder<>() { @Override public WritableFrameChannel get() @@ -214,7 +214,7 @@ public void test_runWithInputChannel() throws Exception new DefaultObjectMapper(), ReadableInput.channel(inputChannel.readable(), FrameReader.create(signature), stagePartition), Function.identity(), - new ResourceHolder() + new ResourceHolder<>() { @Override public WritableFrameChannel get() diff --git a/extensions-core/multi-stage-query/src/test/java/org/apache/druid/msq/test/CalciteSelectJoinQueryMSQTest.java b/extensions-core/multi-stage-query/src/test/java/org/apache/druid/msq/test/CalciteSelectJoinQueryMSQTest.java index 30724dcd4ce0..42ee5bac3528 100644 --- a/extensions-core/multi-stage-query/src/test/java/org/apache/druid/msq/test/CalciteSelectJoinQueryMSQTest.java +++ b/extensions-core/multi-stage-query/src/test/java/org/apache/druid/msq/test/CalciteSelectJoinQueryMSQTest.java @@ -21,11 +21,11 @@ import com.google.common.collect.ImmutableMap; import org.apache.druid.msq.sql.MSQTaskSqlEngine; +import org.apache.druid.query.JoinAlgorithm; import org.apache.druid.sql.calcite.BaseCalciteQueryTest; import org.apache.druid.sql.calcite.CalciteJoinQueryTest; import org.apache.druid.sql.calcite.QueryTestBuilder; import org.apache.druid.sql.calcite.SqlTestFrameworkConfig; -import org.apache.druid.sql.calcite.planner.JoinAlgorithm; import org.apache.druid.sql.calcite.planner.PlannerContext; import java.util.Map; diff --git a/extensions-core/multi-stage-query/src/test/java/org/apache/druid/msq/test/CalciteUnionQueryMSQTest.java b/extensions-core/multi-stage-query/src/test/java/org/apache/druid/msq/test/CalciteUnionQueryMSQTest.java index f0686498786e..a470615fd0d4 100644 --- a/extensions-core/multi-stage-query/src/test/java/org/apache/druid/msq/test/CalciteUnionQueryMSQTest.java +++ b/extensions-core/multi-stage-query/src/test/java/org/apache/druid/msq/test/CalciteUnionQueryMSQTest.java @@ -62,13 +62,12 @@ protected QueryTestBuilder testBuilder() */ @Test @Override - public void testUnionIsUnplannable() + public void testUnionDifferentColumnOrder() { assertQueryIsUnplannable( "SELECT dim2, dim1, m1 FROM foo2 UNION SELECT dim1, dim2, m1 FROM foo", "SQL requires union between two tables and column names queried for each table are different Left: [dim2, dim1, m1], Right: [dim1, dim2, m1]." ); - } @Disabled("Ignored till MSQ can plan UNION ALL with any operand") diff --git a/extensions-core/multi-stage-query/src/test/java/org/apache/druid/msq/test/MSQTestControllerContext.java b/extensions-core/multi-stage-query/src/test/java/org/apache/druid/msq/test/MSQTestControllerContext.java index 22c7cff88477..86051bc4b380 100644 --- a/extensions-core/multi-stage-query/src/test/java/org/apache/druid/msq/test/MSQTestControllerContext.java +++ b/extensions-core/multi-stage-query/src/test/java/org/apache/druid/msq/test/MSQTestControllerContext.java @@ -29,7 +29,6 @@ import com.google.inject.Injector; import org.apache.druid.client.ImmutableSegmentLoadInfo; import org.apache.druid.client.coordinator.CoordinatorClient; -import org.apache.druid.client.indexing.NoopOverlordClient; import org.apache.druid.client.indexing.TaskStatusResponse; import org.apache.druid.common.guava.FutureUtils; import org.apache.druid.indexer.RunnerTaskState; @@ -67,6 +66,7 @@ import org.apache.druid.msq.util.MultiStageQueryContext; import org.apache.druid.query.Query; import org.apache.druid.query.QueryContext; +import org.apache.druid.rpc.indexing.NoopOverlordClient; import org.apache.druid.rpc.indexing.OverlordClient; import org.apache.druid.server.DruidNode; import org.mockito.ArgumentMatchers; diff --git a/extensions-core/multi-stage-query/src/test/java/org/apache/druid/msq/test/MSQTestOverlordServiceClient.java b/extensions-core/multi-stage-query/src/test/java/org/apache/druid/msq/test/MSQTestOverlordServiceClient.java index b35c074fa060..590a086c8e43 100644 --- a/extensions-core/multi-stage-query/src/test/java/org/apache/druid/msq/test/MSQTestOverlordServiceClient.java +++ b/extensions-core/multi-stage-query/src/test/java/org/apache/druid/msq/test/MSQTestOverlordServiceClient.java @@ -28,7 +28,6 @@ import com.google.inject.Injector; import org.apache.calcite.sql.type.SqlTypeName; import org.apache.druid.client.ImmutableSegmentLoadInfo; -import org.apache.druid.client.indexing.NoopOverlordClient; import org.apache.druid.client.indexing.TaskPayloadResponse; import org.apache.druid.client.indexing.TaskStatusResponse; import org.apache.druid.indexer.TaskStatus; @@ -48,6 +47,7 @@ import org.apache.druid.msq.indexing.report.MSQStatusReport; import org.apache.druid.msq.indexing.report.MSQTaskReport; import org.apache.druid.msq.indexing.report.MSQTaskReportPayload; +import org.apache.druid.rpc.indexing.NoopOverlordClient; import org.joda.time.DateTime; import javax.annotation.Nullable; diff --git a/extensions-core/multi-stage-query/src/test/quidem/org.apache.druid.msq.quidem.MSQQuidemTest/msqJoinHint.iq b/extensions-core/multi-stage-query/src/test/quidem/org.apache.druid.msq.quidem.MSQQuidemTest/msqJoinHint.iq new file mode 100644 index 000000000000..d6319fadf9ed --- /dev/null +++ b/extensions-core/multi-stage-query/src/test/quidem/org.apache.druid.msq.quidem.MSQQuidemTest/msqJoinHint.iq @@ -0,0 +1,752 @@ +!use druidtest://?componentSupplier=DrillWindowQueryMSQComponentSupplier +!set outputformat mysql + +select w1.cityName, w2.countryName +from (select cityName, countryName from wikipedia where cityName='New York') w1 +JOIN wikipedia w2 ON w1.cityName = w2.cityName +where w1.cityName='New York'; + +[ { + "stageNumber" : 0, + "definition" : { + "id" : "_0", + "input" : [ { + "type" : "table", + "dataSource" : "wikipedia", + "intervals" : [ "-146136543-09-08T08:23:32.096Z/146140482-04-24T15:36:27.903Z" ], + "filter" : { + "type" : "equals", + "column" : "cityName", + "matchValueType" : "STRING", + "matchValue" : "New York" + }, + "filterFields" : [ "cityName" ] + } ], + "processor" : { + "type" : "scan", + "query" : { + "queryType" : "scan", + "dataSource" : { + "type" : "inputNumber", + "inputNumber" : 0 + }, + "intervals" : { + "type" : "intervals", + "intervals" : [ "-146136543-09-08T08:23:32.096Z/146140482-04-24T15:36:27.903Z" ] + }, + "resultFormat" : "compactedList", + "filter" : { + "type" : "equals", + "column" : "cityName", + "matchValueType" : "STRING", + "matchValue" : "New York" + }, + "columns" : [ "cityName" ], + "context" : { + "scanSignature" : "[{\"name\":\"cityName\",\"type\":\"STRING\"}]", + "sqlInsertSegmentGranularity" : null, + "sqlQueryId" : __SQL_QUERY_ID__ + "sqlStringifyArrays" : false + }, + "columnTypes" : [ "STRING" ], + "granularity" : { + "type" : "all" + }, + "legacy" : false + } + }, + "signature" : [ { + "name" : "__boost", + "type" : "LONG" + }, { + "name" : "cityName", + "type" : "STRING" + } ], + "shuffleSpec" : { + "type" : "maxCount", + "clusterBy" : { + "columns" : [ { + "columnName" : "__boost", + "order" : "ASCENDING" + } ] + }, + "partitions" : 1 + }, + "maxWorkerCount" : 1 + }, + "phase" : "FINISHED", + "workerCount" : 1, + "partitionCount" : 1, + "shuffle" : "globalSort", + "output" : "localStorage", + "startTime" : __TIMESTAMP__ + "duration" : __DURATION__ + "sort" : true +}, { + "stageNumber" : 1, + "definition" : { + "id" : "_1", + "input" : [ { + "type" : "table", + "dataSource" : "wikipedia", + "intervals" : [ "-146136543-09-08T08:23:32.096Z/146140482-04-24T15:36:27.903Z" ] + } ], + "processor" : { + "type" : "scan", + "query" : { + "queryType" : "scan", + "dataSource" : { + "type" : "inputNumber", + "inputNumber" : 0 + }, + "intervals" : { + "type" : "intervals", + "intervals" : [ "-146136543-09-08T08:23:32.096Z/146140482-04-24T15:36:27.903Z" ] + }, + "resultFormat" : "compactedList", + "columns" : [ "cityName", "countryName" ], + "context" : { + "scanSignature" : "[{\"name\":\"cityName\",\"type\":\"STRING\"},{\"name\":\"countryName\",\"type\":\"STRING\"}]", + "sqlInsertSegmentGranularity" : null, + "sqlQueryId" : __SQL_QUERY_ID__ + "sqlStringifyArrays" : false + }, + "columnTypes" : [ "STRING", "STRING" ], + "granularity" : { + "type" : "all" + }, + "legacy" : false + } + }, + "signature" : [ { + "name" : "__boost", + "type" : "LONG" + }, { + "name" : "cityName", + "type" : "STRING" + }, { + "name" : "countryName", + "type" : "STRING" + } ], + "shuffleSpec" : { + "type" : "maxCount", + "clusterBy" : { + "columns" : [ { + "columnName" : "__boost", + "order" : "ASCENDING" + } ] + }, + "partitions" : 1 + }, + "maxWorkerCount" : 1 + }, + "phase" : "FINISHED", + "workerCount" : 1, + "partitionCount" : 1, + "shuffle" : "globalSort", + "output" : "localStorage", + "startTime" : __TIMESTAMP__ + "duration" : __DURATION__ + "sort" : true +}, { + "stageNumber" : 2, + "definition" : { + "id" : "_2", + "input" : [ { + "type" : "stage", + "stage" : 0 + }, { + "type" : "stage", + "stage" : 1 + } ], + "broadcast" : [ 1 ], + "processor" : { + "type" : "scan", + "query" : { + "queryType" : "scan", + "dataSource" : { + "type" : "join", + "left" : { + "type" : "inputNumber", + "inputNumber" : 0 + }, + "right" : { + "type" : "inputNumber", + "inputNumber" : 1 + }, + "rightPrefix" : "j0.", + "condition" : "(\"cityName\" == \"j0.cityName\")", + "joinType" : "INNER" + }, + "intervals" : { + "type" : "intervals", + "intervals" : [ "-146136543-09-08T08:23:32.096Z/146140482-04-24T15:36:27.903Z" ] + }, + "virtualColumns" : [ { + "type" : "expression", + "name" : "v0", + "expression" : "'New York'", + "outputType" : "STRING" + } ], + "resultFormat" : "compactedList", + "columns" : [ "v0", "j0.countryName" ], + "context" : { + "__user" : null, + "finalize" : true, + "maxParseExceptions" : 0, + "scanSignature" : "[{\"name\":\"v0\",\"type\":\"STRING\"},{\"name\":\"j0.countryName\",\"type\":\"STRING\"}]", + "sqlQueryId" : __SQL_QUERY_ID__ + "sqlStringifyArrays" : false, + "windowFunctionOperatorTransformation" : true + }, + "columnTypes" : [ "STRING", "STRING" ], + "granularity" : { + "type" : "all" + }, + "legacy" : false + } + }, + "signature" : [ { + "name" : "__boost", + "type" : "LONG" + }, { + "name" : "v0", + "type" : "STRING" + }, { + "name" : "j0.countryName", + "type" : "STRING" + } ], + "shuffleSpec" : { + "type" : "maxCount", + "clusterBy" : { + "columns" : [ { + "columnName" : "__boost", + "order" : "ASCENDING" + } ] + }, + "partitions" : 1 + }, + "maxWorkerCount" : 1 + }, + "phase" : "FINISHED", + "workerCount" : 1, + "partitionCount" : 1, + "shuffle" : "globalSort", + "output" : "localStorage", + "startTime" : __TIMESTAMP__ + "duration" : __DURATION__ + "sort" : true +} ] +!msqPlan + +select /*+ broadcast */ w1.cityName, w2.countryName +from (select cityName, countryName from wikipedia where cityName='New York') w1 +JOIN wikipedia w2 ON w1.cityName = w2.cityName +where w1.cityName='New York'; + +LogicalJoin:[[broadcast inheritPath:[0, 0]]] + +!hints + +[ { + "stageNumber" : 0, + "definition" : { + "id" : "_0", + "input" : [ { + "type" : "table", + "dataSource" : "wikipedia", + "intervals" : [ "-146136543-09-08T08:23:32.096Z/146140482-04-24T15:36:27.903Z" ], + "filter" : { + "type" : "equals", + "column" : "cityName", + "matchValueType" : "STRING", + "matchValue" : "New York" + }, + "filterFields" : [ "cityName" ] + } ], + "processor" : { + "type" : "scan", + "query" : { + "queryType" : "scan", + "dataSource" : { + "type" : "inputNumber", + "inputNumber" : 0 + }, + "intervals" : { + "type" : "intervals", + "intervals" : [ "-146136543-09-08T08:23:32.096Z/146140482-04-24T15:36:27.903Z" ] + }, + "resultFormat" : "compactedList", + "filter" : { + "type" : "equals", + "column" : "cityName", + "matchValueType" : "STRING", + "matchValue" : "New York" + }, + "columns" : [ "cityName" ], + "context" : { + "scanSignature" : "[{\"name\":\"cityName\",\"type\":\"STRING\"}]", + "sqlInsertSegmentGranularity" : null, + "sqlQueryId" : __SQL_QUERY_ID__ + "sqlStringifyArrays" : false + }, + "columnTypes" : [ "STRING" ], + "granularity" : { + "type" : "all" + }, + "legacy" : false + } + }, + "signature" : [ { + "name" : "__boost", + "type" : "LONG" + }, { + "name" : "cityName", + "type" : "STRING" + } ], + "shuffleSpec" : { + "type" : "maxCount", + "clusterBy" : { + "columns" : [ { + "columnName" : "__boost", + "order" : "ASCENDING" + } ] + }, + "partitions" : 1 + }, + "maxWorkerCount" : 1 + }, + "phase" : "FINISHED", + "workerCount" : 1, + "partitionCount" : 1, + "shuffle" : "globalSort", + "output" : "localStorage", + "startTime" : __TIMESTAMP__ + "duration" : __DURATION__ + "sort" : true +}, { + "stageNumber" : 1, + "definition" : { + "id" : "_1", + "input" : [ { + "type" : "table", + "dataSource" : "wikipedia", + "intervals" : [ "-146136543-09-08T08:23:32.096Z/146140482-04-24T15:36:27.903Z" ] + } ], + "processor" : { + "type" : "scan", + "query" : { + "queryType" : "scan", + "dataSource" : { + "type" : "inputNumber", + "inputNumber" : 0 + }, + "intervals" : { + "type" : "intervals", + "intervals" : [ "-146136543-09-08T08:23:32.096Z/146140482-04-24T15:36:27.903Z" ] + }, + "resultFormat" : "compactedList", + "columns" : [ "cityName", "countryName" ], + "context" : { + "scanSignature" : "[{\"name\":\"cityName\",\"type\":\"STRING\"},{\"name\":\"countryName\",\"type\":\"STRING\"}]", + "sqlInsertSegmentGranularity" : null, + "sqlQueryId" : __SQL_QUERY_ID__ + "sqlStringifyArrays" : false + }, + "columnTypes" : [ "STRING", "STRING" ], + "granularity" : { + "type" : "all" + }, + "legacy" : false + } + }, + "signature" : [ { + "name" : "__boost", + "type" : "LONG" + }, { + "name" : "cityName", + "type" : "STRING" + }, { + "name" : "countryName", + "type" : "STRING" + } ], + "shuffleSpec" : { + "type" : "maxCount", + "clusterBy" : { + "columns" : [ { + "columnName" : "__boost", + "order" : "ASCENDING" + } ] + }, + "partitions" : 1 + }, + "maxWorkerCount" : 1 + }, + "phase" : "FINISHED", + "workerCount" : 1, + "partitionCount" : 1, + "shuffle" : "globalSort", + "output" : "localStorage", + "startTime" : __TIMESTAMP__ + "duration" : __DURATION__ + "sort" : true +}, { + "stageNumber" : 2, + "definition" : { + "id" : "_2", + "input" : [ { + "type" : "stage", + "stage" : 0 + }, { + "type" : "stage", + "stage" : 1 + } ], + "broadcast" : [ 1 ], + "processor" : { + "type" : "scan", + "query" : { + "queryType" : "scan", + "dataSource" : { + "type" : "join", + "left" : { + "type" : "inputNumber", + "inputNumber" : 0 + }, + "right" : { + "type" : "inputNumber", + "inputNumber" : 1 + }, + "rightPrefix" : "j0.", + "condition" : "(\"cityName\" == \"j0.cityName\")", + "joinType" : "INNER" + }, + "intervals" : { + "type" : "intervals", + "intervals" : [ "-146136543-09-08T08:23:32.096Z/146140482-04-24T15:36:27.903Z" ] + }, + "virtualColumns" : [ { + "type" : "expression", + "name" : "v0", + "expression" : "'New York'", + "outputType" : "STRING" + } ], + "resultFormat" : "compactedList", + "columns" : [ "v0", "j0.countryName" ], + "context" : { + "__user" : null, + "finalize" : true, + "maxParseExceptions" : 0, + "scanSignature" : "[{\"name\":\"v0\",\"type\":\"STRING\"},{\"name\":\"j0.countryName\",\"type\":\"STRING\"}]", + "sqlQueryId" : __SQL_QUERY_ID__ + "sqlStringifyArrays" : false, + "windowFunctionOperatorTransformation" : true + }, + "columnTypes" : [ "STRING", "STRING" ], + "granularity" : { + "type" : "all" + }, + "legacy" : false + } + }, + "signature" : [ { + "name" : "__boost", + "type" : "LONG" + }, { + "name" : "v0", + "type" : "STRING" + }, { + "name" : "j0.countryName", + "type" : "STRING" + } ], + "shuffleSpec" : { + "type" : "maxCount", + "clusterBy" : { + "columns" : [ { + "columnName" : "__boost", + "order" : "ASCENDING" + } ] + }, + "partitions" : 1 + }, + "maxWorkerCount" : 1 + }, + "phase" : "FINISHED", + "workerCount" : 1, + "partitionCount" : 1, + "shuffle" : "globalSort", + "output" : "localStorage", + "startTime" : __TIMESTAMP__ + "duration" : __DURATION__ + "sort" : true +} ] +!msqPlan + +select /*+ sort_merge */ w1.cityName, w2.countryName +from (select cityName, countryName from wikipedia where cityName='New York') w1 +JOIN wikipedia w2 ON w1.cityName = w2.cityName +where w1.cityName='New York'; + + +LogicalJoin:[[sort_merge inheritPath:[0, 0]]] + +!hints + +[ { + "stageNumber" : 0, + "definition" : { + "id" : "_0", + "input" : [ { + "type" : "table", + "dataSource" : "wikipedia", + "intervals" : [ "-146136543-09-08T08:23:32.096Z/146140482-04-24T15:36:27.903Z" ], + "filter" : { + "type" : "equals", + "column" : "cityName", + "matchValueType" : "STRING", + "matchValue" : "New York" + }, + "filterFields" : [ "cityName" ] + } ], + "processor" : { + "type" : "scan", + "query" : { + "queryType" : "scan", + "dataSource" : { + "type" : "inputNumber", + "inputNumber" : 0 + }, + "intervals" : { + "type" : "intervals", + "intervals" : [ "-146136543-09-08T08:23:32.096Z/146140482-04-24T15:36:27.903Z" ] + }, + "resultFormat" : "compactedList", + "filter" : { + "type" : "equals", + "column" : "cityName", + "matchValueType" : "STRING", + "matchValue" : "New York" + }, + "columns" : [ "cityName" ], + "context" : { + "scanSignature" : "[{\"name\":\"cityName\",\"type\":\"STRING\"}]", + "sqlInsertSegmentGranularity" : null, + "sqlQueryId" : __SQL_QUERY_ID__ + "sqlStringifyArrays" : false + }, + "columnTypes" : [ "STRING" ], + "granularity" : { + "type" : "all" + }, + "legacy" : false + } + }, + "signature" : [ { + "name" : "cityName", + "type" : "STRING" + }, { + "name" : "__boost", + "type" : "LONG" + } ], + "shuffleSpec" : { + "type" : "hash", + "clusterBy" : { + "columns" : [ { + "columnName" : "cityName", + "order" : "ASCENDING" + } ] + }, + "partitions" : 1 + }, + "maxWorkerCount" : 1 + }, + "phase" : "FINISHED", + "workerCount" : 1, + "partitionCount" : 1, + "shuffle" : "hashLocalSort", + "output" : "localStorage", + "startTime" : __TIMESTAMP__ + "duration" : __DURATION__ + "sort" : true +}, { + "stageNumber" : 1, + "definition" : { + "id" : "_1", + "input" : [ { + "type" : "table", + "dataSource" : "wikipedia", + "intervals" : [ "-146136543-09-08T08:23:32.096Z/146140482-04-24T15:36:27.903Z" ] + } ], + "processor" : { + "type" : "scan", + "query" : { + "queryType" : "scan", + "dataSource" : { + "type" : "inputNumber", + "inputNumber" : 0 + }, + "intervals" : { + "type" : "intervals", + "intervals" : [ "-146136543-09-08T08:23:32.096Z/146140482-04-24T15:36:27.903Z" ] + }, + "resultFormat" : "compactedList", + "columns" : [ "cityName", "countryName" ], + "context" : { + "scanSignature" : "[{\"name\":\"cityName\",\"type\":\"STRING\"},{\"name\":\"countryName\",\"type\":\"STRING\"}]", + "sqlInsertSegmentGranularity" : null, + "sqlQueryId" : __SQL_QUERY_ID__ + "sqlStringifyArrays" : false + }, + "columnTypes" : [ "STRING", "STRING" ], + "granularity" : { + "type" : "all" + }, + "legacy" : false + } + }, + "signature" : [ { + "name" : "cityName", + "type" : "STRING" + }, { + "name" : "__boost", + "type" : "LONG" + }, { + "name" : "countryName", + "type" : "STRING" + } ], + "shuffleSpec" : { + "type" : "hash", + "clusterBy" : { + "columns" : [ { + "columnName" : "cityName", + "order" : "ASCENDING" + } ] + }, + "partitions" : 1 + }, + "maxWorkerCount" : 1 + }, + "phase" : "FINISHED", + "workerCount" : 1, + "partitionCount" : 1, + "shuffle" : "hashLocalSort", + "output" : "localStorage", + "startTime" : __TIMESTAMP__ + "duration" : __DURATION__ + "sort" : true +}, { + "stageNumber" : 2, + "definition" : { + "id" : "_2", + "input" : [ { + "type" : "stage", + "stage" : 0 + }, { + "type" : "stage", + "stage" : 1 + } ], + "processor" : { + "type" : "sortMergeJoin", + "rightPrefix" : "j0.", + "condition" : "(\"cityName\" == \"j0.cityName\")", + "joinType" : "INNER" + }, + "signature" : [ { + "name" : "cityName", + "type" : "STRING" + }, { + "name" : "__boost", + "type" : "LONG" + }, { + "name" : "j0.cityName", + "type" : "STRING" + }, { + "name" : "j0.__boost", + "type" : "LONG" + }, { + "name" : "j0.countryName", + "type" : "STRING" + } ], + "maxWorkerCount" : 1 + }, + "phase" : "FINISHED", + "workerCount" : 1, + "partitionCount" : 1, + "output" : "localStorage", + "startTime" : __TIMESTAMP__ + "duration" : __DURATION__ +}, { + "stageNumber" : 3, + "definition" : { + "id" : "_3", + "input" : [ { + "type" : "stage", + "stage" : 2 + } ], + "processor" : { + "type" : "scan", + "query" : { + "queryType" : "scan", + "dataSource" : { + "type" : "inputNumber", + "inputNumber" : 0 + }, + "intervals" : { + "type" : "intervals", + "intervals" : [ "-146136543-09-08T08:23:32.096Z/146140482-04-24T15:36:27.903Z" ] + }, + "virtualColumns" : [ { + "type" : "expression", + "name" : "v0", + "expression" : "'New York'", + "outputType" : "STRING" + } ], + "resultFormat" : "compactedList", + "columns" : [ "v0", "j0.countryName" ], + "context" : { + "__user" : null, + "finalize" : true, + "maxParseExceptions" : 0, + "scanSignature" : "[{\"name\":\"v0\",\"type\":\"STRING\"},{\"name\":\"j0.countryName\",\"type\":\"STRING\"}]", + "sqlQueryId" : __SQL_QUERY_ID__ + "sqlStringifyArrays" : false, + "windowFunctionOperatorTransformation" : true + }, + "columnTypes" : [ "STRING", "STRING" ], + "granularity" : { + "type" : "all" + }, + "legacy" : false + } + }, + "signature" : [ { + "name" : "__boost", + "type" : "LONG" + }, { + "name" : "v0", + "type" : "STRING" + }, { + "name" : "j0.countryName", + "type" : "STRING" + } ], + "shuffleSpec" : { + "type" : "maxCount", + "clusterBy" : { + "columns" : [ { + "columnName" : "__boost", + "order" : "ASCENDING" + } ] + }, + "partitions" : 1 + }, + "maxWorkerCount" : 1 + }, + "phase" : "FINISHED", + "workerCount" : 1, + "partitionCount" : 1, + "shuffle" : "globalSort", + "output" : "localStorage", + "startTime" : __TIMESTAMP__ + "duration" : __DURATION__ + "sort" : true +} ] +!msqPlan diff --git a/extensions-core/multi-stage-query/src/test/quidem/org.apache.druid.msq.quidem.MSQQuidemTest/msqNestedJoinHint.iq b/extensions-core/multi-stage-query/src/test/quidem/org.apache.druid.msq.quidem.MSQQuidemTest/msqNestedJoinHint.iq new file mode 100644 index 000000000000..acf6240d5467 --- /dev/null +++ b/extensions-core/multi-stage-query/src/test/quidem/org.apache.druid.msq.quidem.MSQQuidemTest/msqNestedJoinHint.iq @@ -0,0 +1,913 @@ +!use druidtest://?componentSupplier=DrillWindowQueryMSQComponentSupplier +!set outputformat mysql + +select w1.cityName, w2.countryName +from +( + select w3.cityName AS cityName, w4.countryName AS countryName from wikipedia w3 LEFT JOIN wikipedia w4 ON w3.regionName = w4.regionName +) w1 +JOIN wikipedia w2 ON w1.cityName = w2.cityName +where w1.cityName='New York'; + +[ { + "stageNumber" : 0, + "definition" : { + "id" : "_0", + "input" : [ { + "type" : "table", + "dataSource" : "wikipedia", + "intervals" : [ "-146136543-09-08T08:23:32.096Z/146140482-04-24T15:36:27.903Z" ] + } ], + "processor" : { + "type" : "scan", + "query" : { + "queryType" : "scan", + "dataSource" : { + "type" : "inputNumber", + "inputNumber" : 0 + }, + "intervals" : { + "type" : "intervals", + "intervals" : [ "-146136543-09-08T08:23:32.096Z/146140482-04-24T15:36:27.903Z" ] + }, + "resultFormat" : "compactedList", + "columns" : [ "regionName" ], + "context" : { + "scanSignature" : "[{\"name\":\"regionName\",\"type\":\"STRING\"}]", + "sqlInsertSegmentGranularity" : null, + "sqlQueryId" : __SQL_QUERY_ID__ + "sqlStringifyArrays" : false + }, + "columnTypes" : [ "STRING" ], + "granularity" : { + "type" : "all" + }, + "legacy" : false + } + }, + "signature" : [ { + "name" : "__boost", + "type" : "LONG" + }, { + "name" : "regionName", + "type" : "STRING" + } ], + "shuffleSpec" : { + "type" : "maxCount", + "clusterBy" : { + "columns" : [ { + "columnName" : "__boost", + "order" : "ASCENDING" + } ] + }, + "partitions" : 1 + }, + "maxWorkerCount" : 1 + }, + "phase" : "FINISHED", + "workerCount" : 1, + "partitionCount" : 1, + "shuffle" : "globalSort", + "output" : "localStorage", + "startTime" : __TIMESTAMP__ + "duration" : __DURATION__ + "sort" : true +}, { + "stageNumber" : 1, + "definition" : { + "id" : "_1", + "input" : [ { + "type" : "table", + "dataSource" : "wikipedia", + "intervals" : [ "-146136543-09-08T08:23:32.096Z/146140482-04-24T15:36:27.903Z" ] + } ], + "processor" : { + "type" : "scan", + "query" : { + "queryType" : "scan", + "dataSource" : { + "type" : "inputNumber", + "inputNumber" : 0 + }, + "intervals" : { + "type" : "intervals", + "intervals" : [ "-146136543-09-08T08:23:32.096Z/146140482-04-24T15:36:27.903Z" ] + }, + "resultFormat" : "compactedList", + "columns" : [ "cityName", "countryName" ], + "context" : { + "scanSignature" : "[{\"name\":\"cityName\",\"type\":\"STRING\"},{\"name\":\"countryName\",\"type\":\"STRING\"}]", + "sqlInsertSegmentGranularity" : null, + "sqlQueryId" : __SQL_QUERY_ID__ + "sqlStringifyArrays" : false + }, + "columnTypes" : [ "STRING", "STRING" ], + "granularity" : { + "type" : "all" + }, + "legacy" : false + } + }, + "signature" : [ { + "name" : "__boost", + "type" : "LONG" + }, { + "name" : "cityName", + "type" : "STRING" + }, { + "name" : "countryName", + "type" : "STRING" + } ], + "shuffleSpec" : { + "type" : "maxCount", + "clusterBy" : { + "columns" : [ { + "columnName" : "__boost", + "order" : "ASCENDING" + } ] + }, + "partitions" : 1 + }, + "maxWorkerCount" : 1 + }, + "phase" : "FINISHED", + "workerCount" : 1, + "partitionCount" : 1, + "shuffle" : "globalSort", + "output" : "localStorage", + "startTime" : __TIMESTAMP__ + "duration" : __DURATION__ + "sort" : true +}, { + "stageNumber" : 2, + "definition" : { + "id" : "_2", + "input" : [ { + "type" : "table", + "dataSource" : "wikipedia", + "intervals" : [ "-146136543-09-08T08:23:32.096Z/146140482-04-24T15:36:27.903Z" ], + "filter" : { + "type" : "equals", + "column" : "cityName", + "matchValueType" : "STRING", + "matchValue" : "New York" + }, + "filterFields" : [ "cityName" ] + }, { + "type" : "stage", + "stage" : 0 + }, { + "type" : "stage", + "stage" : 1 + } ], + "broadcast" : [ 1, 2 ], + "processor" : { + "type" : "scan", + "query" : { + "queryType" : "scan", + "dataSource" : { + "type" : "join", + "left" : { + "type" : "join", + "left" : { + "type" : "inputNumber", + "inputNumber" : 0 + }, + "right" : { + "type" : "inputNumber", + "inputNumber" : 1 + }, + "rightPrefix" : "j0.", + "condition" : "(\"regionName\" == \"j0.regionName\")", + "joinType" : "LEFT" + }, + "right" : { + "type" : "inputNumber", + "inputNumber" : 2 + }, + "rightPrefix" : "_j0.", + "condition" : "(\"cityName\" == \"_j0.cityName\")", + "joinType" : "INNER" + }, + "intervals" : { + "type" : "intervals", + "intervals" : [ "-146136543-09-08T08:23:32.096Z/146140482-04-24T15:36:27.903Z" ] + }, + "virtualColumns" : [ { + "type" : "expression", + "name" : "v0", + "expression" : "'New York'", + "outputType" : "STRING" + } ], + "resultFormat" : "compactedList", + "filter" : { + "type" : "equals", + "column" : "cityName", + "matchValueType" : "STRING", + "matchValue" : "New York" + }, + "columns" : [ "v0", "_j0.countryName" ], + "context" : { + "__user" : null, + "finalize" : true, + "maxParseExceptions" : 0, + "scanSignature" : "[{\"name\":\"v0\",\"type\":\"STRING\"},{\"name\":\"_j0.countryName\",\"type\":\"STRING\"}]", + "sqlQueryId" : __SQL_QUERY_ID__ + "sqlStringifyArrays" : false, + "windowFunctionOperatorTransformation" : true + }, + "columnTypes" : [ "STRING", "STRING" ], + "granularity" : { + "type" : "all" + }, + "legacy" : false + } + }, + "signature" : [ { + "name" : "__boost", + "type" : "LONG" + }, { + "name" : "v0", + "type" : "STRING" + }, { + "name" : "_j0.countryName", + "type" : "STRING" + } ], + "shuffleSpec" : { + "type" : "maxCount", + "clusterBy" : { + "columns" : [ { + "columnName" : "__boost", + "order" : "ASCENDING" + } ] + }, + "partitions" : 1 + }, + "maxWorkerCount" : 1 + }, + "phase" : "FINISHED", + "workerCount" : 1, + "partitionCount" : 1, + "shuffle" : "globalSort", + "output" : "localStorage", + "startTime" : __TIMESTAMP__ + "duration" : __DURATION__ + "sort" : true +} ] +!msqPlan + +select w1.cityName, w2.countryName +from +( + select /*+ sort_merge */ w3.cityName AS cityName, w4.countryName AS countryName from wikipedia w3 LEFT JOIN wikipedia w4 ON w3.regionName = w4.regionName +) w1 +JOIN wikipedia w2 ON w1.cityName = w2.cityName +where w1.cityName='New York'; + +LogicalJoin:[[sort_merge inheritPath:[0]]] + +!hints + +[ { + "stageNumber" : 0, + "definition" : { + "id" : "_0", + "input" : [ { + "type" : "table", + "dataSource" : "wikipedia", + "intervals" : [ "-146136543-09-08T08:23:32.096Z/146140482-04-24T15:36:27.903Z" ] + } ], + "processor" : { + "type" : "scan", + "query" : { + "queryType" : "scan", + "dataSource" : { + "type" : "inputNumber", + "inputNumber" : 0 + }, + "intervals" : { + "type" : "intervals", + "intervals" : [ "-146136543-09-08T08:23:32.096Z/146140482-04-24T15:36:27.903Z" ] + }, + "resultFormat" : "compactedList", + "columns" : [ "cityName", "regionName" ], + "context" : { + "scanSignature" : "[{\"name\":\"cityName\",\"type\":\"STRING\"},{\"name\":\"regionName\",\"type\":\"STRING\"}]", + "sqlInsertSegmentGranularity" : null, + "sqlQueryId" : __SQL_QUERY_ID__ + "sqlStringifyArrays" : false + }, + "columnTypes" : [ "STRING", "STRING" ], + "granularity" : { + "type" : "all" + }, + "legacy" : false + } + }, + "signature" : [ { + "name" : "__boost", + "type" : "LONG" + }, { + "name" : "cityName", + "type" : "STRING" + }, { + "name" : "regionName", + "type" : "STRING" + } ], + "shuffleSpec" : { + "type" : "maxCount", + "clusterBy" : { + "columns" : [ { + "columnName" : "__boost", + "order" : "ASCENDING" + } ] + }, + "partitions" : 1 + }, + "maxWorkerCount" : 1 + }, + "phase" : "FINISHED", + "workerCount" : 1, + "partitionCount" : 1, + "shuffle" : "globalSort", + "output" : "localStorage", + "startTime" : __TIMESTAMP__ + "duration" : __DURATION__ + "sort" : true +}, { + "stageNumber" : 1, + "definition" : { + "id" : "_1", + "input" : [ { + "type" : "table", + "dataSource" : "wikipedia", + "intervals" : [ "-146136543-09-08T08:23:32.096Z/146140482-04-24T15:36:27.903Z" ] + } ], + "processor" : { + "type" : "scan", + "query" : { + "queryType" : "scan", + "dataSource" : { + "type" : "inputNumber", + "inputNumber" : 0 + }, + "intervals" : { + "type" : "intervals", + "intervals" : [ "-146136543-09-08T08:23:32.096Z/146140482-04-24T15:36:27.903Z" ] + }, + "resultFormat" : "compactedList", + "columns" : [ "regionName" ], + "context" : { + "scanSignature" : "[{\"name\":\"regionName\",\"type\":\"STRING\"}]", + "sqlInsertSegmentGranularity" : null, + "sqlQueryId" : __SQL_QUERY_ID__ + "sqlStringifyArrays" : false + }, + "columnTypes" : [ "STRING" ], + "granularity" : { + "type" : "all" + }, + "legacy" : false + } + }, + "signature" : [ { + "name" : "__boost", + "type" : "LONG" + }, { + "name" : "regionName", + "type" : "STRING" + } ], + "shuffleSpec" : { + "type" : "maxCount", + "clusterBy" : { + "columns" : [ { + "columnName" : "__boost", + "order" : "ASCENDING" + } ] + }, + "partitions" : 1 + }, + "maxWorkerCount" : 1 + }, + "phase" : "FINISHED", + "workerCount" : 1, + "partitionCount" : 1, + "shuffle" : "globalSort", + "output" : "localStorage", + "startTime" : __TIMESTAMP__ + "duration" : __DURATION__ + "sort" : true +}, { + "stageNumber" : 2, + "definition" : { + "id" : "_2", + "input" : [ { + "type" : "table", + "dataSource" : "wikipedia", + "intervals" : [ "-146136543-09-08T08:23:32.096Z/146140482-04-24T15:36:27.903Z" ] + } ], + "processor" : { + "type" : "scan", + "query" : { + "queryType" : "scan", + "dataSource" : { + "type" : "inputNumber", + "inputNumber" : 0 + }, + "intervals" : { + "type" : "intervals", + "intervals" : [ "-146136543-09-08T08:23:32.096Z/146140482-04-24T15:36:27.903Z" ] + }, + "resultFormat" : "compactedList", + "columns" : [ "cityName", "countryName" ], + "context" : { + "scanSignature" : "[{\"name\":\"cityName\",\"type\":\"STRING\"},{\"name\":\"countryName\",\"type\":\"STRING\"}]", + "sqlInsertSegmentGranularity" : null, + "sqlQueryId" : __SQL_QUERY_ID__ + "sqlStringifyArrays" : false + }, + "columnTypes" : [ "STRING", "STRING" ], + "granularity" : { + "type" : "all" + }, + "legacy" : false + } + }, + "signature" : [ { + "name" : "__boost", + "type" : "LONG" + }, { + "name" : "cityName", + "type" : "STRING" + }, { + "name" : "countryName", + "type" : "STRING" + } ], + "shuffleSpec" : { + "type" : "maxCount", + "clusterBy" : { + "columns" : [ { + "columnName" : "__boost", + "order" : "ASCENDING" + } ] + }, + "partitions" : 1 + }, + "maxWorkerCount" : 1 + }, + "phase" : "FINISHED", + "workerCount" : 1, + "partitionCount" : 1, + "shuffle" : "globalSort", + "output" : "localStorage", + "startTime" : __TIMESTAMP__ + "duration" : __DURATION__ + "sort" : true +}, { + "stageNumber" : 3, + "definition" : { + "id" : "_3", + "input" : [ { + "type" : "stage", + "stage" : 0 + }, { + "type" : "stage", + "stage" : 1 + }, { + "type" : "stage", + "stage" : 2 + } ], + "broadcast" : [ 1, 2 ], + "processor" : { + "type" : "scan", + "query" : { + "queryType" : "scan", + "dataSource" : { + "type" : "join", + "left" : { + "type" : "join", + "left" : { + "type" : "inputNumber", + "inputNumber" : 0 + }, + "right" : { + "type" : "inputNumber", + "inputNumber" : 1 + }, + "rightPrefix" : "j0.", + "condition" : "(\"regionName\" == \"j0.regionName\")", + "joinType" : "LEFT", + "joinAlgorithm" : "sortMerge" + }, + "right" : { + "type" : "inputNumber", + "inputNumber" : 2 + }, + "rightPrefix" : "_j0.", + "condition" : "(\"cityName\" == \"_j0.cityName\")", + "joinType" : "INNER" + }, + "intervals" : { + "type" : "intervals", + "intervals" : [ "-146136543-09-08T08:23:32.096Z/146140482-04-24T15:36:27.903Z" ] + }, + "virtualColumns" : [ { + "type" : "expression", + "name" : "v0", + "expression" : "'New York'", + "outputType" : "STRING" + } ], + "resultFormat" : "compactedList", + "filter" : { + "type" : "equals", + "column" : "cityName", + "matchValueType" : "STRING", + "matchValue" : "New York" + }, + "columns" : [ "v0", "_j0.countryName" ], + "context" : { + "__user" : null, + "finalize" : true, + "maxParseExceptions" : 0, + "scanSignature" : "[{\"name\":\"v0\",\"type\":\"STRING\"},{\"name\":\"_j0.countryName\",\"type\":\"STRING\"}]", + "sqlQueryId" : __SQL_QUERY_ID__ + "sqlStringifyArrays" : false, + "windowFunctionOperatorTransformation" : true + }, + "columnTypes" : [ "STRING", "STRING" ], + "granularity" : { + "type" : "all" + }, + "legacy" : false + } + }, + "signature" : [ { + "name" : "__boost", + "type" : "LONG" + }, { + "name" : "v0", + "type" : "STRING" + }, { + "name" : "_j0.countryName", + "type" : "STRING" + } ], + "shuffleSpec" : { + "type" : "maxCount", + "clusterBy" : { + "columns" : [ { + "columnName" : "__boost", + "order" : "ASCENDING" + } ] + }, + "partitions" : 1 + }, + "maxWorkerCount" : 1 + }, + "phase" : "FINISHED", + "workerCount" : 1, + "partitionCount" : 1, + "shuffle" : "globalSort", + "output" : "localStorage", + "startTime" : __TIMESTAMP__ + "duration" : __DURATION__ + "sort" : true +} ] +!msqPlan + +select /*+ sort_merge */ w1.cityName, w2.countryName +from +( + select /*+ broadcast */ w3.cityName AS cityName, w4.countryName AS countryName from wikipedia w3 LEFT JOIN wikipedia w4 ON w3.regionName = w4.regionName +) w1 +JOIN wikipedia w2 ON w1.cityName = w2.cityName +where w1.cityName='New York'; + +[ { + "stageNumber" : 0, + "definition" : { + "id" : "_0", + "input" : [ { + "type" : "table", + "dataSource" : "wikipedia", + "intervals" : [ "-146136543-09-08T08:23:32.096Z/146140482-04-24T15:36:27.903Z" ] + } ], + "processor" : { + "type" : "scan", + "query" : { + "queryType" : "scan", + "dataSource" : { + "type" : "inputNumber", + "inputNumber" : 0 + }, + "intervals" : { + "type" : "intervals", + "intervals" : [ "-146136543-09-08T08:23:32.096Z/146140482-04-24T15:36:27.903Z" ] + }, + "resultFormat" : "compactedList", + "columns" : [ "regionName" ], + "context" : { + "scanSignature" : "[{\"name\":\"regionName\",\"type\":\"STRING\"}]", + "sqlInsertSegmentGranularity" : null, + "sqlQueryId" : __SQL_QUERY_ID__ + "sqlStringifyArrays" : false + }, + "columnTypes" : [ "STRING" ], + "granularity" : { + "type" : "all" + }, + "legacy" : false + } + }, + "signature" : [ { + "name" : "__boost", + "type" : "LONG" + }, { + "name" : "regionName", + "type" : "STRING" + } ], + "shuffleSpec" : { + "type" : "maxCount", + "clusterBy" : { + "columns" : [ { + "columnName" : "__boost", + "order" : "ASCENDING" + } ] + }, + "partitions" : 1 + }, + "maxWorkerCount" : 1 + }, + "phase" : "FINISHED", + "workerCount" : 1, + "partitionCount" : 1, + "shuffle" : "globalSort", + "output" : "localStorage", + "startTime" : __TIMESTAMP__ + "duration" : __DURATION__ + "sort" : true +}, { + "stageNumber" : 1, + "definition" : { + "id" : "_1", + "input" : [ { + "type" : "table", + "dataSource" : "wikipedia", + "intervals" : [ "-146136543-09-08T08:23:32.096Z/146140482-04-24T15:36:27.903Z" ] + }, { + "type" : "stage", + "stage" : 0 + } ], + "broadcast" : [ 1 ], + "processor" : { + "type" : "scan", + "query" : { + "queryType" : "scan", + "dataSource" : { + "type" : "join", + "left" : { + "type" : "inputNumber", + "inputNumber" : 0 + }, + "right" : { + "type" : "inputNumber", + "inputNumber" : 1 + }, + "rightPrefix" : "j0.", + "condition" : "(\"regionName\" == \"j0.regionName\")", + "joinType" : "LEFT" + }, + "intervals" : { + "type" : "intervals", + "intervals" : [ "-146136543-09-08T08:23:32.096Z/146140482-04-24T15:36:27.903Z" ] + }, + "resultFormat" : "compactedList", + "columns" : [ "cityName" ], + "context" : { + "scanSignature" : "[{\"name\":\"cityName\",\"type\":\"STRING\"}]", + "sqlInsertSegmentGranularity" : null, + "sqlQueryId" : __SQL_QUERY_ID__ + "sqlStringifyArrays" : false + }, + "columnTypes" : [ "STRING" ], + "granularity" : { + "type" : "all" + }, + "legacy" : false + } + }, + "signature" : [ { + "name" : "cityName", + "type" : "STRING" + }, { + "name" : "__boost", + "type" : "LONG" + } ], + "shuffleSpec" : { + "type" : "hash", + "clusterBy" : { + "columns" : [ { + "columnName" : "cityName", + "order" : "ASCENDING" + } ] + }, + "partitions" : 1 + }, + "maxWorkerCount" : 1 + }, + "phase" : "FINISHED", + "workerCount" : 1, + "partitionCount" : 1, + "shuffle" : "hashLocalSort", + "output" : "localStorage", + "startTime" : __TIMESTAMP__ + "duration" : __DURATION__ + "sort" : true +}, { + "stageNumber" : 2, + "definition" : { + "id" : "_2", + "input" : [ { + "type" : "table", + "dataSource" : "wikipedia", + "intervals" : [ "-146136543-09-08T08:23:32.096Z/146140482-04-24T15:36:27.903Z" ] + } ], + "processor" : { + "type" : "scan", + "query" : { + "queryType" : "scan", + "dataSource" : { + "type" : "inputNumber", + "inputNumber" : 0 + }, + "intervals" : { + "type" : "intervals", + "intervals" : [ "-146136543-09-08T08:23:32.096Z/146140482-04-24T15:36:27.903Z" ] + }, + "resultFormat" : "compactedList", + "columns" : [ "cityName", "countryName" ], + "context" : { + "scanSignature" : "[{\"name\":\"cityName\",\"type\":\"STRING\"},{\"name\":\"countryName\",\"type\":\"STRING\"}]", + "sqlInsertSegmentGranularity" : null, + "sqlQueryId" : __SQL_QUERY_ID__ + "sqlStringifyArrays" : false + }, + "columnTypes" : [ "STRING", "STRING" ], + "granularity" : { + "type" : "all" + }, + "legacy" : false + } + }, + "signature" : [ { + "name" : "cityName", + "type" : "STRING" + }, { + "name" : "__boost", + "type" : "LONG" + }, { + "name" : "countryName", + "type" : "STRING" + } ], + "shuffleSpec" : { + "type" : "hash", + "clusterBy" : { + "columns" : [ { + "columnName" : "cityName", + "order" : "ASCENDING" + } ] + }, + "partitions" : 1 + }, + "maxWorkerCount" : 1 + }, + "phase" : "FINISHED", + "workerCount" : 1, + "partitionCount" : 1, + "shuffle" : "hashLocalSort", + "output" : "localStorage", + "startTime" : __TIMESTAMP__ + "duration" : __DURATION__ + "sort" : true +}, { + "stageNumber" : 3, + "definition" : { + "id" : "_3", + "input" : [ { + "type" : "stage", + "stage" : 1 + }, { + "type" : "stage", + "stage" : 2 + } ], + "processor" : { + "type" : "sortMergeJoin", + "rightPrefix" : "_j0.", + "condition" : "(\"cityName\" == \"_j0.cityName\")", + "joinType" : "INNER" + }, + "signature" : [ { + "name" : "cityName", + "type" : "STRING" + }, { + "name" : "__boost", + "type" : "LONG" + }, { + "name" : "_j0.cityName", + "type" : "STRING" + }, { + "name" : "_j0.__boost", + "type" : "LONG" + }, { + "name" : "_j0.countryName", + "type" : "STRING" + } ], + "maxWorkerCount" : 1 + }, + "phase" : "FINISHED", + "workerCount" : 1, + "partitionCount" : 1, + "output" : "localStorage", + "startTime" : __TIMESTAMP__ + "duration" : __DURATION__ +}, { + "stageNumber" : 4, + "definition" : { + "id" : "_4", + "input" : [ { + "type" : "stage", + "stage" : 3 + } ], + "processor" : { + "type" : "scan", + "query" : { + "queryType" : "scan", + "dataSource" : { + "type" : "inputNumber", + "inputNumber" : 0 + }, + "intervals" : { + "type" : "intervals", + "intervals" : [ "-146136543-09-08T08:23:32.096Z/146140482-04-24T15:36:27.903Z" ] + }, + "virtualColumns" : [ { + "type" : "expression", + "name" : "v0", + "expression" : "'New York'", + "outputType" : "STRING" + } ], + "resultFormat" : "compactedList", + "filter" : { + "type" : "equals", + "column" : "cityName", + "matchValueType" : "STRING", + "matchValue" : "New York" + }, + "columns" : [ "v0", "_j0.countryName" ], + "context" : { + "__user" : null, + "finalize" : true, + "maxParseExceptions" : 0, + "scanSignature" : "[{\"name\":\"v0\",\"type\":\"STRING\"},{\"name\":\"_j0.countryName\",\"type\":\"STRING\"}]", + "sqlQueryId" : __SQL_QUERY_ID__ + "sqlStringifyArrays" : false, + "windowFunctionOperatorTransformation" : true + }, + "columnTypes" : [ "STRING", "STRING" ], + "granularity" : { + "type" : "all" + }, + "legacy" : false + } + }, + "signature" : [ { + "name" : "__boost", + "type" : "LONG" + }, { + "name" : "v0", + "type" : "STRING" + }, { + "name" : "_j0.countryName", + "type" : "STRING" + } ], + "shuffleSpec" : { + "type" : "maxCount", + "clusterBy" : { + "columns" : [ { + "columnName" : "__boost", + "order" : "ASCENDING" + } ] + }, + "partitions" : 1 + }, + "maxWorkerCount" : 1 + }, + "phase" : "FINISHED", + "workerCount" : 1, + "partitionCount" : 1, + "shuffle" : "globalSort", + "output" : "localStorage", + "startTime" : __TIMESTAMP__ + "duration" : __DURATION__ + "sort" : true +} ] +!msqPlan diff --git a/extensions-core/mysql-metadata-storage/pom.xml b/extensions-core/mysql-metadata-storage/pom.xml index 57f5ab4f12aa..103bb005f4ea 100644 --- a/extensions-core/mysql-metadata-storage/pom.xml +++ b/extensions-core/mysql-metadata-storage/pom.xml @@ -78,11 +78,6 @@ jackson-databind provided - - com.google.inject.extensions - guice-multibindings - provided - org.apache.commons commons-dbcp2 diff --git a/extensions-core/orc-extensions/src/main/java/org/apache/druid/data/input/orc/OrcReader.java b/extensions-core/orc-extensions/src/main/java/org/apache/druid/data/input/orc/OrcReader.java index 3f4c0386d46a..de49086cc01b 100644 --- a/extensions-core/orc-extensions/src/main/java/org/apache/druid/data/input/orc/OrcReader.java +++ b/extensions-core/orc-extensions/src/main/java/org/apache/druid/data/input/orc/OrcReader.java @@ -105,7 +105,7 @@ protected CloseableIterator intermediateRowIterator() throws IOExcept final RecordReader batchReader = reader.rows(reader.options()); final OrcMapredRecordReader recordReader = new OrcMapredRecordReader<>(batchReader, schema); closer.register(recordReader::close); - return new CloseableIterator() + return new CloseableIterator<>() { final NullWritable key = recordReader.createKey(); OrcStruct value = null; diff --git a/extensions-core/parquet-extensions/src/main/java/org/apache/druid/data/input/parquet/ParquetReader.java b/extensions-core/parquet-extensions/src/main/java/org/apache/druid/data/input/parquet/ParquetReader.java index 0bc5f374a0cf..26b5d6d4774b 100644 --- a/extensions-core/parquet-extensions/src/main/java/org/apache/druid/data/input/parquet/ParquetReader.java +++ b/extensions-core/parquet-extensions/src/main/java/org/apache/druid/data/input/parquet/ParquetReader.java @@ -101,7 +101,7 @@ protected CloseableIterator intermediateRowIterator() throws IOException Thread.currentThread().setContextClassLoader(currentClassLoader); } - return new CloseableIterator() + return new CloseableIterator<>() { Group value = null; diff --git a/extensions-core/postgresql-metadata-storage/pom.xml b/extensions-core/postgresql-metadata-storage/pom.xml index 9c4159e36e81..fafd54869abb 100644 --- a/extensions-core/postgresql-metadata-storage/pom.xml +++ b/extensions-core/postgresql-metadata-storage/pom.xml @@ -76,11 +76,7 @@ jackson-databind provided - - com.google.inject.extensions - guice-multibindings - provided - + org.apache.commons commons-dbcp2 diff --git a/extensions-core/postgresql-metadata-storage/src/main/java/org/apache/druid/metadata/storage/postgresql/PostgreSQLConnector.java b/extensions-core/postgresql-metadata-storage/src/main/java/org/apache/druid/metadata/storage/postgresql/PostgreSQLConnector.java index f0b9ba0b4a0b..765e12aa39a8 100644 --- a/extensions-core/postgresql-metadata-storage/src/main/java/org/apache/druid/metadata/storage/postgresql/PostgreSQLConnector.java +++ b/extensions-core/postgresql-metadata-storage/src/main/java/org/apache/druid/metadata/storage/postgresql/PostgreSQLConnector.java @@ -187,7 +187,7 @@ public Void insertOrUpdate( ) { return getDBI().withHandle( - new HandleCallback() + new HandleCallback<>() { @Override public Void withHandle(Handle handle) throws Exception diff --git a/extensions-core/s3-extensions/pom.xml b/extensions-core/s3-extensions/pom.xml index b50d9c01a9e0..04df8b58f96f 100644 --- a/extensions-core/s3-extensions/pom.xml +++ b/extensions-core/s3-extensions/pom.xml @@ -87,11 +87,6 @@ jackson-core provided - - com.google.inject.extensions - guice-multibindings - provided - org.apache.commons commons-lang3 @@ -159,6 +154,50 @@ equalsverifier test + + org.testcontainers + testcontainers + test + + + org.testcontainers + junit-jupiter + test + + + org.junit.jupiter + junit-jupiter-api + test + + + org.testcontainers + minio + 1.19.5 + test + + + + local + + true + + + + + org.apache.maven.plugins + maven-surefire-plugin + + !requires-dockerd + + + + + + + ci + + + diff --git a/extensions-core/s3-extensions/src/main/java/org/apache/druid/storage/s3/S3DataSegmentMover.java b/extensions-core/s3-extensions/src/main/java/org/apache/druid/storage/s3/S3DataSegmentMover.java index d7c92a309f12..770fe356e498 100644 --- a/extensions-core/s3-extensions/src/main/java/org/apache/druid/storage/s3/S3DataSegmentMover.java +++ b/extensions-core/s3-extensions/src/main/java/org/apache/druid/storage/s3/S3DataSegmentMover.java @@ -100,7 +100,7 @@ public DataSegment move(DataSegment segment, Map targetLoadSpec) .putAll( Maps.filterKeys( loadSpec, - new Predicate() + new Predicate<>() { @Override public boolean apply(String input) diff --git a/extensions-core/s3-extensions/src/main/java/org/apache/druid/storage/s3/S3Utils.java b/extensions-core/s3-extensions/src/main/java/org/apache/druid/storage/s3/S3Utils.java index ecea31c94f74..b299d4f9dd80 100644 --- a/extensions-core/s3-extensions/src/main/java/org/apache/druid/storage/s3/S3Utils.java +++ b/extensions-core/s3-extensions/src/main/java/org/apache/druid/storage/s3/S3Utils.java @@ -70,7 +70,7 @@ public class S3Utils */ public static final String ERROR_ENTITY_TOO_LARGE = "EntityTooLarge"; - public static final Predicate S3RETRY = new Predicate() + public static final Predicate S3RETRY = new Predicate<>() { @Override public boolean apply(Throwable e) diff --git a/extensions-core/s3-extensions/src/main/java/org/apache/druid/storage/s3/ServerSideEncryptingAmazonS3.java b/extensions-core/s3-extensions/src/main/java/org/apache/druid/storage/s3/ServerSideEncryptingAmazonS3.java index 0fb93a208337..31120ba883c4 100644 --- a/extensions-core/s3-extensions/src/main/java/org/apache/druid/storage/s3/ServerSideEncryptingAmazonS3.java +++ b/extensions-core/s3-extensions/src/main/java/org/apache/druid/storage/s3/ServerSideEncryptingAmazonS3.java @@ -72,6 +72,11 @@ public ServerSideEncryptingAmazonS3(AmazonS3 amazonS3, ServerSideEncryption serv this.serverSideEncryption = serverSideEncryption; } + public AmazonS3 getAmazonS3() + { + return amazonS3; + } + public boolean doesObjectExist(String bucket, String objectName) { try { diff --git a/extensions-core/s3-extensions/src/main/java/org/apache/druid/storage/s3/output/S3StorageConnector.java b/extensions-core/s3-extensions/src/main/java/org/apache/druid/storage/s3/output/S3StorageConnector.java index 8eb391a24d39..99d0207bc282 100644 --- a/extensions-core/s3-extensions/src/main/java/org/apache/druid/storage/s3/output/S3StorageConnector.java +++ b/extensions-core/s3-extensions/src/main/java/org/apache/druid/storage/s3/output/S3StorageConnector.java @@ -57,8 +57,8 @@ public class S3StorageConnector extends ChunkingStorageConnector buildInputParams(Str builder.maxRetry(config.getMaxRetry()); builder.retryCondition(S3Utils.S3RETRY); builder.objectSupplier((start, end) -> new GetObjectRequest(config.getBucket(), objectPath(path)).withRange(start, end - 1)); - builder.objectOpenFunction(new ObjectOpenFunction() + builder.objectOpenFunction(new ObjectOpenFunction<>() { @Override public InputStream open(GetObjectRequest object) diff --git a/extensions-core/s3-extensions/src/test/java/org/apache/druid/data/input/s3/S3InputSourceTest.java b/extensions-core/s3-extensions/src/test/java/org/apache/druid/data/input/s3/S3InputSourceTest.java index bbb1b294abf7..4f14364e7222 100644 --- a/extensions-core/s3-extensions/src/test/java/org/apache/druid/data/input/s3/S3InputSourceTest.java +++ b/extensions-core/s3-extensions/src/test/java/org/apache/druid/data/input/s3/S3InputSourceTest.java @@ -1352,8 +1352,8 @@ public List getJacksonModules() // See https://github.com/FasterXML/jackson-databind/issues/962. return ImmutableList.of( new SimpleModule() - .addDeserializer(AmazonS3.class, new ItemDeserializer()) - .addDeserializer(AmazonS3ClientBuilder.class, new ItemDeserializer()) + .addDeserializer(AmazonS3.class, new ItemDeserializer<>()) + .addDeserializer(AmazonS3ClientBuilder.class, new ItemDeserializer<>()) ); } diff --git a/extensions-core/s3-extensions/src/test/java/org/apache/druid/storage/s3/output/S3StorageConnectorTest.java b/extensions-core/s3-extensions/src/test/java/org/apache/druid/storage/s3/output/S3StorageConnectorTest.java index 68eaca1c42a3..56df24d21152 100644 --- a/extensions-core/s3-extensions/src/test/java/org/apache/druid/storage/s3/output/S3StorageConnectorTest.java +++ b/extensions-core/s3-extensions/src/test/java/org/apache/druid/storage/s3/output/S3StorageConnectorTest.java @@ -19,160 +19,147 @@ package org.apache.druid.storage.s3.output; +import com.amazonaws.ClientConfigurationFactory; +import com.amazonaws.auth.AWSCredentials; +import com.amazonaws.auth.BasicAWSCredentials; +import com.amazonaws.client.builder.AwsClientBuilder; +import com.amazonaws.internal.StaticCredentialsProvider; import com.amazonaws.services.s3.AmazonS3Client; +import com.amazonaws.services.s3.AmazonS3ClientBuilder; import com.amazonaws.services.s3.model.AmazonS3Exception; -import com.amazonaws.services.s3.model.DeleteObjectsRequest; -import com.amazonaws.services.s3.model.GetObjectMetadataRequest; -import com.amazonaws.services.s3.model.GetObjectRequest; -import com.amazonaws.services.s3.model.ListObjectsV2Request; -import com.amazonaws.services.s3.model.ListObjectsV2Result; -import com.amazonaws.services.s3.model.ObjectMetadata; -import com.amazonaws.services.s3.model.S3Object; -import com.amazonaws.services.s3.model.S3ObjectSummary; +import com.amazonaws.services.s3.model.CreateBucketRequest; import com.google.common.collect.ImmutableList; import com.google.common.collect.Lists; import org.apache.druid.java.util.common.HumanReadableBytes; +import org.apache.druid.java.util.common.StringUtils; import org.apache.druid.java.util.metrics.StubServiceEmitter; import org.apache.druid.query.DruidProcessingConfigTest; import org.apache.druid.storage.StorageConnector; -import org.apache.druid.storage.s3.NoopServerSideEncryption; import org.apache.druid.storage.s3.ServerSideEncryptingAmazonS3; -import org.easymock.Capture; -import org.easymock.EasyMock; -import org.hamcrest.CoreMatchers; -import org.hamcrest.MatcherAssert; -import org.junit.Assert; -import org.junit.Before; -import org.junit.Rule; -import org.junit.Test; -import org.junit.internal.matchers.ThrowableCauseMatcher; -import org.junit.rules.TemporaryFolder; - -import java.io.BufferedReader; -import java.io.ByteArrayInputStream; +import org.junit.jupiter.api.Assertions; +import org.junit.jupiter.api.BeforeEach; +import org.junit.jupiter.api.Tag; +import org.junit.jupiter.api.Test; +import org.junit.jupiter.api.io.TempDir; +import org.testcontainers.containers.MinIOContainer; +import org.testcontainers.junit.jupiter.Container; +import org.testcontainers.junit.jupiter.Testcontainers; +import org.testcontainers.utility.DockerImageName; + +import java.io.File; import java.io.IOException; import java.io.InputStream; -import java.io.InputStreamReader; +import java.io.OutputStream; import java.nio.charset.StandardCharsets; -import java.util.Collections; +import java.nio.file.Files; +import java.nio.file.Path; +import java.nio.file.Paths; +import java.util.ArrayList; import java.util.List; -import java.util.stream.Collectors; +import java.util.UUID; +import static org.apache.druid.storage.s3.output.S3StorageConnector.JOINER; + +@Testcontainers +@Tag("requires-dockerd") public class S3StorageConnectorTest { - private static final String BUCKET = "BUCKET"; + private static final String BUCKET = "testbucket"; private static final String PREFIX = "P/R/E/F/I/X"; public static final String TEST_FILE = "test.csv"; - - private final AmazonS3Client s3Client = EasyMock.createMock(AmazonS3Client.class); - private final ServerSideEncryptingAmazonS3 service = new ServerSideEncryptingAmazonS3( - s3Client, - new NoopServerSideEncryption() - ); - private final ListObjectsV2Result testResult = EasyMock.createMock(ListObjectsV2Result.class); - - @Rule - public TemporaryFolder temporaryFolder = new TemporaryFolder(); + @Container + private static final MinIOContainer MINIO = MinioUtil.createContainer(); + @TempDir + public static File temporaryFolder; + private ServerSideEncryptingAmazonS3 s3Client; private StorageConnector storageConnector; - @Before - public void setup() + @BeforeEach + public void setup() throws IOException { - try { - S3OutputConfig s3OutputConfig = new S3OutputConfig( - BUCKET, - PREFIX, - temporaryFolder.newFolder(), - null, - null, - true - ); - storageConnector = new S3StorageConnector(s3OutputConfig, service, new S3UploadManager( - s3OutputConfig, - new S3ExportConfig("tempDir", new HumanReadableBytes("5MiB"), 1, null), - new DruidProcessingConfigTest.MockRuntimeInfo(10, 0, 0), - new StubServiceEmitter())); - } - catch (IOException e) { - throw new RuntimeException(e); + s3Client = MinioUtil.createS3Client(MINIO); + if (!s3Client.getAmazonS3().doesBucketExistV2(BUCKET)) { + s3Client.getAmazonS3().createBucket(new CreateBucketRequest(BUCKET)); } + + S3OutputConfig s3OutputConfig = new S3OutputConfig( + BUCKET, + PREFIX, + Files.createDirectory(Paths.get(temporaryFolder.getAbsolutePath(), UUID.randomUUID().toString())).toFile(), + null, + null, + true + ); + storageConnector = new S3StorageConnector( + s3OutputConfig, + s3Client, + new S3UploadManager( + s3OutputConfig, + new S3ExportConfig("tempDir", new HumanReadableBytes("5MiB"), 1, null), + new DruidProcessingConfigTest.MockRuntimeInfo(10, 0, 0), + new StubServiceEmitter() + ) + ); } @Test public void pathExists_yes() throws IOException { - final Capture request = Capture.newInstance(); - EasyMock.reset(s3Client); - EasyMock.expect(s3Client.getObjectMetadata(EasyMock.capture(request))) - .andReturn(new ObjectMetadata()); - EasyMock.replay(s3Client); - Assert.assertTrue(storageConnector.pathExists(TEST_FILE)); - Assert.assertEquals(BUCKET, request.getValue().getBucketName()); - Assert.assertEquals(PREFIX + "/" + TEST_FILE, request.getValue().getKey()); - EasyMock.verify(s3Client); + s3Client.putObject( + BUCKET, + JOINER.join(PREFIX, TEST_FILE), + Files.createFile(Path.of(temporaryFolder.toPath().toString(), TEST_FILE)).toFile() + ); + Assertions.assertTrue(storageConnector.pathExists(TEST_FILE)); } @Test public void pathExists_notFound() throws IOException { - final Capture request = Capture.newInstance(); - final AmazonS3Exception e = new AmazonS3Exception("not found"); - e.setStatusCode(404); - - EasyMock.reset(s3Client); - EasyMock.expect(s3Client.getObjectMetadata(EasyMock.capture(request))) - .andThrow(e); - EasyMock.replay(s3Client); - Assert.assertFalse(storageConnector.pathExists(TEST_FILE)); - Assert.assertEquals(BUCKET, request.getValue().getBucketName()); - Assert.assertEquals(PREFIX + "/" + TEST_FILE, request.getValue().getKey()); - EasyMock.verify(s3Client); + Assertions.assertFalse(storageConnector.pathExists(UUID.randomUUID().toString())); } @Test - public void pathExists_error() + public void pathExists_error() throws IOException { - final Capture request = Capture.newInstance(); - final AmazonS3Exception e = new AmazonS3Exception("not found"); - e.setStatusCode(403); - - EasyMock.reset(s3Client); - EasyMock.expect(s3Client.getObjectMetadata(EasyMock.capture(request))) - .andThrow(e); - EasyMock.replay(s3Client); - final IOException e2 = Assert.assertThrows( + S3OutputConfig s3OutputConfig = new S3OutputConfig( + BUCKET, + PREFIX, + Files.createDirectory(Paths.get(temporaryFolder.getAbsolutePath(), UUID.randomUUID().toString())).toFile(), + null, + null, + true + ); + StorageConnector unauthorizedStorageConnector = new S3StorageConnector( + s3OutputConfig, + MinioUtil.createUnauthorizedS3Client(MINIO), + new S3UploadManager( + s3OutputConfig, + new S3ExportConfig("tempDir", new HumanReadableBytes("5MiB"), 1, null), + new DruidProcessingConfigTest.MockRuntimeInfo(10, 0, 0), + new StubServiceEmitter() + ) + ); + final IOException e2 = Assertions.assertThrows( IOException.class, - () -> storageConnector.pathExists(TEST_FILE) + () -> unauthorizedStorageConnector.pathExists(TEST_FILE) ); - Assert.assertEquals(BUCKET, request.getValue().getBucketName()); - Assert.assertEquals(PREFIX + "/" + TEST_FILE, request.getValue().getKey()); - MatcherAssert.assertThat(e2, ThrowableCauseMatcher.hasCause(CoreMatchers.instanceOf(AmazonS3Exception.class))); - EasyMock.verify(s3Client); + Assertions.assertEquals(AmazonS3Exception.class, e2.getCause().getClass()); + AmazonS3Exception amazonS3Exception = (AmazonS3Exception) e2.getCause(); + Assertions.assertEquals(403, amazonS3Exception.getStatusCode()); } @Test public void pathRead() throws IOException { - EasyMock.reset(s3Client); - ObjectMetadata objectMetadata = new ObjectMetadata(); - long contentLength = "test".getBytes(StandardCharsets.UTF_8).length; - objectMetadata.setContentLength(contentLength); - S3Object s3Object = new S3Object(); - s3Object.setObjectContent(new ByteArrayInputStream("test".getBytes(StandardCharsets.UTF_8))); - EasyMock.expect(s3Client.getObjectMetadata(EasyMock.anyObject())).andReturn(objectMetadata); - EasyMock.expect(s3Client.getObject( - new GetObjectRequest(BUCKET, PREFIX + "/" + TEST_FILE).withRange(0, contentLength - 1)) - ).andReturn(s3Object); - EasyMock.replay(s3Client); - - String readText = new BufferedReader( - new InputStreamReader(storageConnector.read(TEST_FILE), StandardCharsets.UTF_8)) - .lines() - .collect(Collectors.joining("\n")); - - Assert.assertEquals("test", readText); - EasyMock.reset(s3Client); + try (OutputStream outputStream = storageConnector.write("readWrite1")) { + outputStream.write("test".getBytes(StandardCharsets.UTF_8)); + } + try (InputStream inputStream = storageConnector.read("readWrite1")) { + byte[] bytes = inputStream.readAllBytes(); + Assertions.assertEquals("test", new String(bytes, StandardCharsets.UTF_8)); + } } @Test @@ -180,139 +167,175 @@ public void testReadRange() throws IOException { String data = "test"; + try (OutputStream outputStream = storageConnector.write("readWrite2")) { + outputStream.write(data.getBytes(StandardCharsets.UTF_8)); + } + // non empty reads for (int start = 0; start < data.length(); start++) { for (int length = 1; length <= data.length() - start; length++) { String dataQueried = data.substring(start, start + length); - EasyMock.reset(s3Client); - S3Object s3Object = new S3Object(); - s3Object.setObjectContent( - new ByteArrayInputStream(dataQueried.getBytes(StandardCharsets.UTF_8)) - ); - EasyMock.expect( - s3Client.getObject( - new GetObjectRequest(BUCKET, PREFIX + "/" + TEST_FILE).withRange(start, start + length - 1) - ) - ).andReturn(s3Object); - EasyMock.replay(s3Client); - - InputStream is = storageConnector.readRange(TEST_FILE, start, length); - byte[] dataBytes = new byte[length]; - Assert.assertEquals(length, is.read(dataBytes)); - Assert.assertEquals(-1, is.read()); // reading further produces no data - Assert.assertEquals(dataQueried, new String(dataBytes, StandardCharsets.UTF_8)); - EasyMock.reset(s3Client); + try (InputStream inputStream = storageConnector.readRange("readWrite2", start, length)) { + byte[] bytes = inputStream.readAllBytes(); + Assertions.assertEquals(dataQueried, new String(bytes, StandardCharsets.UTF_8)); + } } } // empty read - EasyMock.reset(s3Client); - S3Object s3Object = new S3Object(); - s3Object.setObjectContent( - new ByteArrayInputStream("".getBytes(StandardCharsets.UTF_8)) - ); - EasyMock.expect( - s3Client.getObject( - new GetObjectRequest(BUCKET, PREFIX + "/" + TEST_FILE).withRange(0, -1) - ) - ).andReturn(s3Object); - EasyMock.replay(s3Client); - - InputStream is = storageConnector.readRange(TEST_FILE, 0, 0); - byte[] dataBytes = new byte[0]; - Assert.assertEquals(is.read(dataBytes), -1); - Assert.assertEquals("", new String(dataBytes, StandardCharsets.UTF_8)); - EasyMock.reset(s3Client); + try (InputStream inputStream = storageConnector.readRange("readWrite2", 0, 0)) { + byte[] bytes = inputStream.readAllBytes(); + Assertions.assertEquals("", new String(bytes, StandardCharsets.UTF_8)); + } } @Test public void testDeleteSinglePath() throws IOException { - EasyMock.reset(s3Client); - s3Client.deleteObject(BUCKET, PREFIX + "/" + TEST_FILE); - EasyMock.expectLastCall(); - storageConnector.deleteFile(TEST_FILE); - EasyMock.reset(s3Client); + String deleteFolderName = UUID.randomUUID().toString(); + try (OutputStream outputStream = storageConnector.write(StringUtils.format("%s/deleteSingle", deleteFolderName))) { + outputStream.write("delete".getBytes(StandardCharsets.UTF_8)); + } + + ArrayList listResult = new ArrayList<>(); + storageConnector.listDir(deleteFolderName + "/").forEachRemaining(listResult::add); + Assertions.assertEquals(1, listResult.size()); + Assertions.assertEquals("deleteSingle", listResult.get(0)); + storageConnector.deleteFile(StringUtils.format("%s/deleteSingle", deleteFolderName)); + + listResult.clear(); + storageConnector.listDir(deleteFolderName + "/").forEachRemaining(listResult::add); + Assertions.assertEquals(0, listResult.size()); } @Test public void testDeleteMultiplePaths() throws IOException { - EasyMock.reset(s3Client); - String testFile2 = "file2"; - DeleteObjectsRequest deleteObjectsRequest = new DeleteObjectsRequest(BUCKET); - deleteObjectsRequest.withKeys(PREFIX + "/" + TEST_FILE, PREFIX + "/" + testFile2); - Capture capturedArgument = EasyMock.newCapture(); - - EasyMock.expect(s3Client.deleteObjects(EasyMock.capture(capturedArgument))).andReturn(null).once(); - EasyMock.replay(s3Client); - storageConnector.deleteFiles(Lists.newArrayList(TEST_FILE, testFile2)); - - Assert.assertEquals( - convertDeleteObjectsRequestToString(deleteObjectsRequest), - convertDeleteObjectsRequestToString(capturedArgument.getValue()) - ); - EasyMock.reset(s3Client); + String deleteFolderName = UUID.randomUUID().toString(); + try (OutputStream outputStream = storageConnector.write(StringUtils.format("%s/deleteFirst", deleteFolderName))) { + outputStream.write("first".getBytes(StandardCharsets.UTF_8)); + } + try (OutputStream outputStream = storageConnector.write(StringUtils.format("%s/deleteSecond", deleteFolderName))) { + outputStream.write("second".getBytes(StandardCharsets.UTF_8)); + } + + ArrayList listResult = new ArrayList<>(); + storageConnector.listDir(deleteFolderName + "/").forEachRemaining(listResult::add); + Assertions.assertEquals(2, listResult.size()); + Assertions.assertEquals("deleteFirst", listResult.get(0)); + Assertions.assertEquals("deleteSecond", listResult.get(1)); + + storageConnector.deleteFiles(ImmutableList.of( + StringUtils.format("%s/deleteFirst", deleteFolderName), + StringUtils.format("%s/deleteSecond", deleteFolderName) + )); + + listResult.clear(); + storageConnector.listDir(deleteFolderName + "/").forEachRemaining(listResult::add); + Assertions.assertEquals(0, listResult.size()); } @Test public void testPathDeleteRecursively() throws IOException { - EasyMock.reset(s3Client, testResult); - - S3ObjectSummary s3ObjectSummary = new S3ObjectSummary(); - s3ObjectSummary.setBucketName(BUCKET); - s3ObjectSummary.setKey(PREFIX + "/test/" + TEST_FILE); - s3ObjectSummary.setSize(1); - EasyMock.expect(s3Client.listObjectsV2((ListObjectsV2Request) EasyMock.anyObject())) - .andReturn(testResult); - - EasyMock.expect(testResult.getBucketName()).andReturn("123").anyTimes(); - EasyMock.expect(testResult.getObjectSummaries()).andReturn(Collections.singletonList(s3ObjectSummary)).anyTimes(); - EasyMock.expect(testResult.isTruncated()).andReturn(false).times(1); - EasyMock.expect(testResult.getNextContinuationToken()).andReturn(null); - - Capture capturedArgument = EasyMock.newCapture(); - EasyMock.expect(s3Client.deleteObjects(EasyMock.and( - EasyMock.capture(capturedArgument), - EasyMock.isA(DeleteObjectsRequest.class) - ))).andReturn(null); - EasyMock.replay(s3Client, testResult); - - storageConnector.deleteRecursively("test"); - - Assert.assertEquals(1, capturedArgument.getValue().getKeys().size()); - Assert.assertEquals(PREFIX + "/test/" + TEST_FILE, capturedArgument.getValue().getKeys().get(0).getKey()); - EasyMock.reset(s3Client, testResult); + String deleteFolderName = UUID.randomUUID().toString(); + try (OutputStream outputStream = storageConnector.write(StringUtils.format("%s/deleteFirst", deleteFolderName))) { + outputStream.write("first".getBytes(StandardCharsets.UTF_8)); + } + try (OutputStream outputStream = storageConnector.write(StringUtils.format("%s/inner/deleteSecond", deleteFolderName))) { + outputStream.write("second".getBytes(StandardCharsets.UTF_8)); + } + + ArrayList listResult = new ArrayList<>(); + storageConnector.listDir(deleteFolderName + "/").forEachRemaining(listResult::add); + Assertions.assertEquals(2, listResult.size()); + Assertions.assertEquals("deleteFirst", listResult.get(0)); + Assertions.assertEquals("inner/deleteSecond", listResult.get(1)); + + storageConnector.deleteRecursively(deleteFolderName); + + listResult.clear(); + storageConnector.listDir(deleteFolderName + "/").forEachRemaining(listResult::add); + Assertions.assertEquals(0, listResult.size()); } @Test public void testListDir() throws IOException { - EasyMock.reset(s3Client, testResult); - - S3ObjectSummary s3ObjectSummary = new S3ObjectSummary(); - s3ObjectSummary.setBucketName(BUCKET); - s3ObjectSummary.setKey(PREFIX + "/test/" + TEST_FILE); - s3ObjectSummary.setSize(1); - - EasyMock.expect(testResult.getObjectSummaries()).andReturn(Collections.singletonList(s3ObjectSummary)).times(2); - EasyMock.expect(testResult.isTruncated()).andReturn(false); - EasyMock.expect(testResult.getNextContinuationToken()).andReturn(null); - EasyMock.expect(s3Client.listObjectsV2((ListObjectsV2Request) EasyMock.anyObject())) - .andReturn(testResult); - EasyMock.replay(s3Client, testResult); - - List listDirResult = Lists.newArrayList(storageConnector.listDir("test/")); - Assert.assertEquals(ImmutableList.of(TEST_FILE), listDirResult); + String listFolderName = UUID.randomUUID().toString(); + try (OutputStream outputStream = storageConnector.write(StringUtils.format("%s/listFirst", listFolderName))) { + outputStream.write("first".getBytes(StandardCharsets.UTF_8)); + } + List listDirResult = Lists.newArrayList(storageConnector.listDir(listFolderName + "/")); + Assertions.assertEquals(ImmutableList.of("listFirst"), listDirResult); } - private String convertDeleteObjectsRequestToString(DeleteObjectsRequest deleteObjectsRequest) + // adapted from apache iceberg tests + private static class MinioUtil { - return deleteObjectsRequest.getKeys() - .stream() - .map(keyVersion -> keyVersion.getKey() + keyVersion.getVersion()) - .collect( - Collectors.joining()); + private MinioUtil() + { + } + + public static MinIOContainer createContainer() + { + return createContainer(null); + } + + public static MinIOContainer createContainer(AWSCredentials credentials) + { + MinIOContainer container = new MinIOContainer(DockerImageName.parse("minio/minio:latest")); + + // this enables virtual-host-style requests. see + // https://github.com/minio/minio/tree/master/docs/config#domain + container.withEnv("MINIO_DOMAIN", "localhost"); + + if (credentials != null) { + container.withUserName(credentials.getAWSAccessKeyId()); + container.withPassword(credentials.getAWSSecretKey()); + } + + return container; + } + + public static ServerSideEncryptingAmazonS3 createS3Client(MinIOContainer container) + { + final AmazonS3ClientBuilder amazonS3ClientBuilder = AmazonS3Client + .builder() + .withEndpointConfiguration( + new AwsClientBuilder.EndpointConfiguration( + container.getS3URL(), + "us-east-1" + ) + ) + .withCredentials(new StaticCredentialsProvider( + new BasicAWSCredentials(container.getUserName(), container.getPassword()))) + .withClientConfiguration(new ClientConfigurationFactory().getConfig()) + .withPathStyleAccessEnabled(true); // OSX won't resolve subdomains + + return ServerSideEncryptingAmazonS3.builder() + .setAmazonS3ClientBuilder(amazonS3ClientBuilder) + .build(); + } + + public static ServerSideEncryptingAmazonS3 createUnauthorizedS3Client(MinIOContainer container) + { + final AmazonS3ClientBuilder amazonS3ClientBuilder = AmazonS3Client + .builder() + .withEndpointConfiguration( + new AwsClientBuilder.EndpointConfiguration( + container.getS3URL(), + "us-east-1" + ) + ) + .withCredentials(new StaticCredentialsProvider( + new BasicAWSCredentials(container.getUserName(), "wrong"))) + .withClientConfiguration(new ClientConfigurationFactory().getConfig()) + .withPathStyleAccessEnabled(true); // OSX won't resolve subdomains + + return ServerSideEncryptingAmazonS3.builder() + .setAmazonS3ClientBuilder(amazonS3ClientBuilder) + .build(); + } } } diff --git a/indexing-hadoop/src/main/java/org/apache/druid/indexer/DetermineHashedPartitionsJob.java b/indexing-hadoop/src/main/java/org/apache/druid/indexer/DetermineHashedPartitionsJob.java index 1d0c3c4b14fc..2328552efb7b 100644 --- a/indexing-hadoop/src/main/java/org/apache/druid/indexer/DetermineHashedPartitionsJob.java +++ b/indexing-hadoop/src/main/java/org/apache/druid/indexer/DetermineHashedPartitionsJob.java @@ -159,9 +159,7 @@ public boolean run() } List intervals = HadoopDruidIndexerConfig.JSON_MAPPER.readValue( Utils.openInputStream(groupByJob, intervalInfoPath), - new TypeReference>() - { - } + new TypeReference<>() {} ); config.setGranularitySpec( new UniformGranularitySpec( diff --git a/indexing-hadoop/src/main/java/org/apache/druid/indexer/DeterminePartitionsJob.java b/indexing-hadoop/src/main/java/org/apache/druid/indexer/DeterminePartitionsJob.java index d7e4eb2e10dc..3efb6c5bec7e 100644 --- a/indexing-hadoop/src/main/java/org/apache/druid/indexer/DeterminePartitionsJob.java +++ b/indexing-hadoop/src/main/java/org/apache/druid/indexer/DeterminePartitionsJob.java @@ -272,9 +272,7 @@ public boolean run() } if (Utils.exists(dimSelectionJob, fileSystem, partitionInfoPath)) { List specs = HadoopDruidIndexerConfig.JSON_MAPPER.readValue( - Utils.openInputStream(dimSelectionJob, partitionInfoPath), new TypeReference>() - { - } + Utils.openInputStream(dimSelectionJob, partitionInfoPath), new TypeReference<>() {} ); List actualSpecs = Lists.newArrayListWithExpectedSize(specs.size()); @@ -619,7 +617,7 @@ protected abstract void innerReduce( private static Iterable combineRows(Iterable input) { - final Comparator> dimsComparator = new Comparator>() + final Comparator> dimsComparator = new Comparator<>() { @Override public int compare(List o1, List o2) diff --git a/indexing-hadoop/src/main/java/org/apache/druid/indexer/HadoopDruidIndexerConfig.java b/indexing-hadoop/src/main/java/org/apache/druid/indexer/HadoopDruidIndexerConfig.java index 246a068e4609..bb830afd244f 100644 --- a/indexing-hadoop/src/main/java/org/apache/druid/indexer/HadoopDruidIndexerConfig.java +++ b/indexing-hadoop/src/main/java/org/apache/druid/indexer/HadoopDruidIndexerConfig.java @@ -488,7 +488,7 @@ Optional> getAllBuckets() return FunctionalIterable .create(specs) .transform( - new Function() + new Function<>() { int i = 0; diff --git a/indexing-hadoop/src/main/java/org/apache/druid/indexer/JobHelper.java b/indexing-hadoop/src/main/java/org/apache/druid/indexer/JobHelper.java index 6b1b84d02d7a..7639ba66b5f6 100644 --- a/indexing-hadoop/src/main/java/org/apache/druid/indexer/JobHelper.java +++ b/indexing-hadoop/src/main/java/org/apache/druid/indexer/JobHelper.java @@ -179,7 +179,7 @@ public static void setupClasspath( public static Predicate shouldRetryPredicate() { - return new Predicate() + return new Predicate<>() { @Override public boolean apply(Throwable input) diff --git a/indexing-hadoop/src/main/java/org/apache/druid/indexer/Utils.java b/indexing-hadoop/src/main/java/org/apache/druid/indexer/Utils.java index 19dfe3093a52..79ed14ad961c 100644 --- a/indexing-hadoop/src/main/java/org/apache/druid/indexer/Utils.java +++ b/indexing-hadoop/src/main/java/org/apache/druid/indexer/Utils.java @@ -232,9 +232,7 @@ private static void checkAppSuccessFromYarnRMOnce( log.info("App status response from YARN RM: " + res.getContentAsString()); Map respMap = HadoopDruidIndexerConfig.JSON_MAPPER.readValue( res.getContentAsString(), - new TypeReference>() - { - } + new TypeReference<>() {} ); Map appMap = (Map) respMap.get("app"); diff --git a/indexing-hadoop/src/main/java/org/apache/druid/indexer/hadoop/DatasourceInputFormat.java b/indexing-hadoop/src/main/java/org/apache/druid/indexer/hadoop/DatasourceInputFormat.java index 0937c19bd569..e6d39284c5af 100644 --- a/indexing-hadoop/src/main/java/org/apache/druid/indexer/hadoop/DatasourceInputFormat.java +++ b/indexing-hadoop/src/main/java/org/apache/druid/indexer/hadoop/DatasourceInputFormat.java @@ -139,7 +139,7 @@ public RecordReader createRecordReader( return new DatasourceRecordReader(); } - private Supplier supplier = new Supplier() + private Supplier supplier = new Supplier<>() { @Override public org.apache.hadoop.mapred.InputFormat get() @@ -253,7 +253,7 @@ public static List getDataSources(final Configuration conf) throws IOExc return HadoopDruidIndexerConfig.JSON_MAPPER.readValue( currentDatasources, - new TypeReference>() {} + new TypeReference<>() {} ); } @@ -286,7 +286,7 @@ public static List getSegments(final Configuration conf, fi { return HadoopDruidIndexerConfig.JSON_MAPPER.readValue( conf.get(StringUtils.format("%s.%s", CONF_SEGMENTS, dataSource)), - new TypeReference>() {} + new TypeReference<>() {} ); } diff --git a/indexing-hadoop/src/main/java/org/apache/druid/indexer/hadoop/DatasourceInputSplit.java b/indexing-hadoop/src/main/java/org/apache/druid/indexer/hadoop/DatasourceInputSplit.java index e842adeed1a7..a2cd03e2a22e 100644 --- a/indexing-hadoop/src/main/java/org/apache/druid/indexer/hadoop/DatasourceInputSplit.java +++ b/indexing-hadoop/src/main/java/org/apache/druid/indexer/hadoop/DatasourceInputSplit.java @@ -91,9 +91,7 @@ public void readFields(DataInput in) throws IOException in.readFully(buf); segments = HadoopDruidIndexerConfig.JSON_MAPPER.readValue( buf, - new TypeReference>() - { - } + new TypeReference<>() {} ); locations = new String[in.readInt()]; for (int i = 0; i < locations.length; i++) { diff --git a/indexing-hadoop/src/main/java/org/apache/druid/indexer/hadoop/DatasourceRecordReader.java b/indexing-hadoop/src/main/java/org/apache/druid/indexer/hadoop/DatasourceRecordReader.java index 5ef56583f7c6..b604937b1b01 100644 --- a/indexing-hadoop/src/main/java/org/apache/druid/indexer/hadoop/DatasourceRecordReader.java +++ b/indexing-hadoop/src/main/java/org/apache/druid/indexer/hadoop/DatasourceRecordReader.java @@ -100,7 +100,7 @@ public void initialize(InputSplit split, final TaskAttemptContext context) throw List adapters = Lists.transform( segments, - new Function() + new Function<>() { @Override public WindowedCursorFactory apply(WindowedDataSegment segment) @@ -205,7 +205,7 @@ public SegmentReader( Sequence rows = Sequences.concat( Iterables.transform( cursorFactories, - new Function>() + new Function<>() { @Nullable @Override @@ -247,7 +247,7 @@ public Sequence apply(WindowedCursorFactory windowed) @Override public Iterator iterator() { - return new Iterator() + return new Iterator<>() { @Override public boolean hasNext() diff --git a/indexing-hadoop/src/main/java/org/apache/druid/indexer/hadoop/FSSpideringIterator.java b/indexing-hadoop/src/main/java/org/apache/druid/indexer/hadoop/FSSpideringIterator.java index 9c76cdd6c55a..8355b354989e 100644 --- a/indexing-hadoop/src/main/java/org/apache/druid/indexer/hadoop/FSSpideringIterator.java +++ b/indexing-hadoop/src/main/java/org/apache/druid/indexer/hadoop/FSSpideringIterator.java @@ -50,7 +50,7 @@ public static FSSpideringIterator spiderPathPropagateExceptions(FileSystem fs, P public static Iterable spiderIterable(final FileSystem fs, final Path path) { - return new Iterable() + return new Iterable<>() { @Override public Iterator iterator() diff --git a/indexing-hadoop/src/test/java/org/apache/druid/indexer/IndexGeneratorJobTest.java b/indexing-hadoop/src/test/java/org/apache/druid/indexer/IndexGeneratorJobTest.java index 241746ca58d6..a53850704e65 100644 --- a/indexing-hadoop/src/test/java/org/apache/druid/indexer/IndexGeneratorJobTest.java +++ b/indexing-hadoop/src/test/java/org/apache/druid/indexer/IndexGeneratorJobTest.java @@ -491,7 +491,7 @@ public void setUp() throws Exception dataFile = temporaryFolder.newFile(); tmpDir = temporaryFolder.newFolder(); - HashMap inputSpec = new HashMap(); + HashMap inputSpec = new HashMap<>(); inputSpec.put("paths", dataFile.getCanonicalPath()); inputSpec.put("type", "static"); if (inputFormatName != null) { diff --git a/indexing-hadoop/src/test/java/org/apache/druid/indexer/JobHelperTest.java b/indexing-hadoop/src/test/java/org/apache/druid/indexer/JobHelperTest.java index 530c0f657a51..d7825aa01478 100644 --- a/indexing-hadoop/src/test/java/org/apache/druid/indexer/JobHelperTest.java +++ b/indexing-hadoop/src/test/java/org/apache/druid/indexer/JobHelperTest.java @@ -311,7 +311,7 @@ public void progress() private static class HadoopDruidIndexerConfigSpy extends HadoopDruidIndexerConfig { - private Map jobProperties = new HashMap(); + private Map jobProperties = new HashMap<>(); public HadoopDruidIndexerConfigSpy(HadoopDruidIndexerConfig delegate) { diff --git a/indexing-hadoop/src/test/java/org/apache/druid/indexer/hadoop/DatasourceInputFormatTest.java b/indexing-hadoop/src/test/java/org/apache/druid/indexer/hadoop/DatasourceInputFormatTest.java index 2da6b210e87e..562ba7f19abc 100644 --- a/indexing-hadoop/src/test/java/org/apache/druid/indexer/hadoop/DatasourceInputFormatTest.java +++ b/indexing-hadoop/src/test/java/org/apache/druid/indexer/hadoop/DatasourceInputFormatTest.java @@ -213,7 +213,7 @@ private static T populateConfiguration( return conf; } - private Supplier testFormatter = new Supplier() + private Supplier testFormatter = new Supplier<>() { @Override public InputFormat get() diff --git a/indexing-hadoop/src/test/java/org/apache/druid/indexer/hadoop/FSSpideringIteratorTest.java b/indexing-hadoop/src/test/java/org/apache/druid/indexer/hadoop/FSSpideringIteratorTest.java index 896ee8460741..f028eeeeec32 100644 --- a/indexing-hadoop/src/test/java/org/apache/druid/indexer/hadoop/FSSpideringIteratorTest.java +++ b/indexing-hadoop/src/test/java/org/apache/druid/indexer/hadoop/FSSpideringIteratorTest.java @@ -63,7 +63,7 @@ public void testNonEmptyDirs() FileSystem.getLocal(new Configuration()), new Path(baseDir.toString()) ), - new Function() + new Function<>() { @Override public String apply(@Nullable FileStatus input) @@ -111,7 +111,7 @@ public void testEmptyDirs() FileSystem.getLocal(new Configuration()), new Path(baseDir.toString()) ), - new Function() + new Function<>() { @Override public String apply(@Nullable FileStatus input) diff --git a/indexing-service/pom.xml b/indexing-service/pom.xml index 412568ea88c9..dd28de400cc2 100644 --- a/indexing-service/pom.xml +++ b/indexing-service/pom.xml @@ -88,11 +88,7 @@ com.fasterxml.jackson.core jackson-databind - - com.google.inject.extensions - guice-multibindings - provided - + javax.ws.rs jsr311-api diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/CheckPointDataSourceMetadataAction.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/CheckPointDataSourceMetadataAction.java index e319dc242660..23fce6740c5d 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/CheckPointDataSourceMetadataAction.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/CheckPointDataSourceMetadataAction.java @@ -106,9 +106,7 @@ public SeekableStreamDataSourceMetadata getCheckpointMetadata() @Override public TypeReference getReturnTypeReference() { - return new TypeReference() - { - }; + return new TypeReference<>() {}; } @Override diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/LocalTaskActionClient.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/LocalTaskActionClient.java index 1d0059335ed1..81ff137ed179 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/LocalTaskActionClient.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/LocalTaskActionClient.java @@ -69,6 +69,7 @@ private R performAction(TaskAction taskAction) return result; } catch (Throwable t) { + log.error(t, "Failed to perform action[%s]", taskAction); throw new RuntimeException(t); } } diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/LockListAction.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/LockListAction.java index c60c57407984..cc7c2907fd18 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/LockListAction.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/LockListAction.java @@ -30,7 +30,7 @@ public class LockListAction implements TaskAction> @Override public TypeReference> getReturnTypeReference() { - return new TypeReference>() {}; + return new TypeReference<>() {}; } @Override diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/LockReleaseAction.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/LockReleaseAction.java index 32974117cc7b..63f97bae69c7 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/LockReleaseAction.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/LockReleaseAction.java @@ -46,7 +46,7 @@ public Interval getInterval() @Override public TypeReference getReturnTypeReference() { - return new TypeReference() {}; + return new TypeReference<>() {}; } @Override diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/MarkSegmentsAsUnusedAction.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/MarkSegmentsAsUnusedAction.java index 35a8ed3e35c0..a3341022f625 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/MarkSegmentsAsUnusedAction.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/MarkSegmentsAsUnusedAction.java @@ -55,9 +55,7 @@ public Interval getInterval() @Override public TypeReference getReturnTypeReference() { - return new TypeReference() - { - }; + return new TypeReference<>() {}; } @Override diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/ResetDataSourceMetadataAction.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/ResetDataSourceMetadataAction.java index 42259b381830..1310b0f69acf 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/ResetDataSourceMetadataAction.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/ResetDataSourceMetadataAction.java @@ -53,9 +53,7 @@ public DataSourceMetadata getResetMetadata() @Override public TypeReference getReturnTypeReference() { - return new TypeReference() - { - }; + return new TypeReference<>() {}; } @Override diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/RetrieveSegmentsByIdAction.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/RetrieveSegmentsByIdAction.java index a4ca90ac62ed..e20096745a7b 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/RetrieveSegmentsByIdAction.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/RetrieveSegmentsByIdAction.java @@ -62,9 +62,7 @@ public Set getSegmentIds() @Override public TypeReference> getReturnTypeReference() { - return new TypeReference>() - { - }; + return new TypeReference<>() {}; } @Override diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/RetrieveUnusedSegmentsAction.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/RetrieveUnusedSegmentsAction.java index fb58328b3d76..a43540cb98ec 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/RetrieveUnusedSegmentsAction.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/RetrieveUnusedSegmentsAction.java @@ -91,7 +91,7 @@ public DateTime getMaxUsedStatusLastUpdatedTime() @Override public TypeReference> getReturnTypeReference() { - return new TypeReference>() {}; + return new TypeReference<>() {}; } @Override diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/RetrieveUpgradedFromSegmentIdsAction.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/RetrieveUpgradedFromSegmentIdsAction.java index 4f54131dee6f..2f86f568c450 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/RetrieveUpgradedFromSegmentIdsAction.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/RetrieveUpgradedFromSegmentIdsAction.java @@ -59,9 +59,7 @@ public Set getSegmentIds() @Override public TypeReference getReturnTypeReference() { - return new TypeReference() - { - }; + return new TypeReference<>() {}; } @Override diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/RetrieveUpgradedToSegmentIdsAction.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/RetrieveUpgradedToSegmentIdsAction.java index 256a663a1a47..ab2547ecb7ef 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/RetrieveUpgradedToSegmentIdsAction.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/RetrieveUpgradedToSegmentIdsAction.java @@ -64,9 +64,7 @@ public Set getSegmentIds() @Override public TypeReference getReturnTypeReference() { - return new TypeReference() - { - }; + return new TypeReference<>() {}; } @Override diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/RetrieveUsedSegmentsAction.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/RetrieveUsedSegmentsAction.java index 473976efd1ea..51bec50a600e 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/RetrieveUsedSegmentsAction.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/RetrieveUsedSegmentsAction.java @@ -107,7 +107,7 @@ public Segments getVisibility() @Override public TypeReference> getReturnTypeReference() { - return new TypeReference>() {}; + return new TypeReference<>() {}; } @Override diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/SegmentAllocateAction.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/SegmentAllocateAction.java index 902dad5dd879..c9581318e711 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/SegmentAllocateAction.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/SegmentAllocateAction.java @@ -184,9 +184,7 @@ public TaskLockType getTaskLockType() @Override public TypeReference getReturnTypeReference() { - return new TypeReference() - { - }; + return new TypeReference<>() {}; } @Override diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/SegmentLockAcquireAction.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/SegmentLockAcquireAction.java index e17af3847ca8..ac922a6aae25 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/SegmentLockAcquireAction.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/SegmentLockAcquireAction.java @@ -93,9 +93,7 @@ public long getTimeoutMs() @Override public TypeReference getReturnTypeReference() { - return new TypeReference() - { - }; + return new TypeReference<>() {}; } @Override diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/SegmentLockTryAcquireAction.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/SegmentLockTryAcquireAction.java index c5084eb61cda..b54004bd3574 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/SegmentLockTryAcquireAction.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/SegmentLockTryAcquireAction.java @@ -86,9 +86,7 @@ public Set getPartitionIds() @Override public TypeReference> getReturnTypeReference() { - return new TypeReference>() - { - }; + return new TypeReference<>() {}; } @Override diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/SegmentMetadataUpdateAction.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/SegmentMetadataUpdateAction.java index b2a5ee5843be..af3f535422b6 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/SegmentMetadataUpdateAction.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/SegmentMetadataUpdateAction.java @@ -56,7 +56,7 @@ public Set getSegments() @Override public TypeReference getReturnTypeReference() { - return new TypeReference() {}; + return new TypeReference<>() {}; } @Override diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/SegmentNukeAction.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/SegmentNukeAction.java index 326907c8f0a5..84445d3968a5 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/SegmentNukeAction.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/SegmentNukeAction.java @@ -56,9 +56,7 @@ public Set getSegments() @Override public TypeReference getReturnTypeReference() { - return new TypeReference() - { - }; + return new TypeReference<>() {}; } @Override diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/SegmentTransactionalAppendAction.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/SegmentTransactionalAppendAction.java index 2326176885b3..ea3a4fd36d2a 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/SegmentTransactionalAppendAction.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/SegmentTransactionalAppendAction.java @@ -132,9 +132,7 @@ public SegmentSchemaMapping getSegmentSchemaMapping() @Override public TypeReference getReturnTypeReference() { - return new TypeReference() - { - }; + return new TypeReference<>() {}; } @Override diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/SegmentTransactionalInsertAction.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/SegmentTransactionalInsertAction.java index e8dd472cf31d..fb988e78e5d4 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/SegmentTransactionalInsertAction.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/SegmentTransactionalInsertAction.java @@ -169,9 +169,7 @@ public SegmentSchemaMapping getSegmentSchemaMapping() @Override public TypeReference getReturnTypeReference() { - return new TypeReference() - { - }; + return new TypeReference<>() {}; } /** diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/SegmentTransactionalReplaceAction.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/SegmentTransactionalReplaceAction.java index 572f4aa3f284..6546ed80d9d9 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/SegmentTransactionalReplaceAction.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/SegmentTransactionalReplaceAction.java @@ -105,9 +105,7 @@ public SegmentSchemaMapping getSegmentSchemaMapping() @Override public TypeReference getReturnTypeReference() { - return new TypeReference() - { - }; + return new TypeReference<>() {}; } /** diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/TimeChunkLockAcquireAction.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/TimeChunkLockAcquireAction.java index 42f4e40d0e64..1b9f60353a6f 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/TimeChunkLockAcquireAction.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/TimeChunkLockAcquireAction.java @@ -76,9 +76,7 @@ public long getTimeoutMs() @Override public TypeReference getReturnTypeReference() { - return new TypeReference() - { - }; + return new TypeReference<>() {}; } @Override diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/TimeChunkLockTryAcquireAction.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/TimeChunkLockTryAcquireAction.java index b2dc6e3d5e62..5bbb7999c61c 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/TimeChunkLockTryAcquireAction.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/TimeChunkLockTryAcquireAction.java @@ -65,9 +65,7 @@ public Interval getInterval() @Override public TypeReference getReturnTypeReference() { - return new TypeReference() - { - }; + return new TypeReference<>() {}; } @Override diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/UpdateLocationAction.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/UpdateLocationAction.java index 2c7c265a7f40..6bce838fc5c0 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/UpdateLocationAction.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/UpdateLocationAction.java @@ -48,9 +48,7 @@ public TaskLocation getTaskLocation() @Override public TypeReference getReturnTypeReference() { - return new TypeReference() - { - }; + return new TypeReference<>() {}; } @Override diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/UpdateStatusAction.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/UpdateStatusAction.java index d02020acb9d0..e5170c79eac2 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/UpdateStatusAction.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/actions/UpdateStatusAction.java @@ -68,9 +68,7 @@ public TaskStatus getStatusFull() @Override public TypeReference getReturnTypeReference() { - return new TypeReference() - { - }; + return new TypeReference<>() {}; } @Override diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/AbstractBatchIndexTask.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/AbstractBatchIndexTask.java index 0f124389399e..185e6e014c69 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/AbstractBatchIndexTask.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/AbstractBatchIndexTask.java @@ -639,7 +639,7 @@ public static Function, Set> addCompactionStateToS : new ClientCompactionTaskTransformSpec(ingestionSpec.getDataSchema().getTransformSpec().getFilter()).asMap(toolbox.getJsonMapper()); List metricsSpec = ingestionSpec.getDataSchema().getAggregators() == null ? null - : toolbox.getJsonMapper().convertValue(ingestionSpec.getDataSchema().getAggregators(), new TypeReference>() {}); + : toolbox.getJsonMapper().convertValue(ingestionSpec.getDataSchema().getAggregators(), new TypeReference<>() {}); return CompactionState.addCompactionStateToSegments( tuningConfig.getPartitionsSpec(), diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/HadoopIndexTask.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/HadoopIndexTask.java index 81721cabb44b..319c1a7ab691 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/HadoopIndexTask.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/HadoopIndexTask.java @@ -921,9 +921,7 @@ public String[] runTask(String[] args) throws Exception public static class HadoopRenameSegmentIndexFilesRunner { TypeReference> LIST_DATA_SEGMENT_AND_INDEX_ZIP_FILE_PATH = - new TypeReference>() - { - }; + new TypeReference<>() {}; public void runTask(String[] args) throws Exception { @@ -963,9 +961,7 @@ public void runTask(String[] args) throws Exception public static class HadoopIndexerGeneratorCleanupRunner { TypeReference> LIST_DATA_SEGMENT_AND_INDEX_ZIP_FILE_PATH = - new TypeReference>() - { - }; + new TypeReference<>() {}; public void runTask(String[] args) throws Exception { diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/HadoopTask.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/HadoopTask.java index 201024540e16..a239025ee05d 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/HadoopTask.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/HadoopTask.java @@ -81,7 +81,7 @@ public List getHadoopDependencyCoordinates() // This is only used for classpath isolation in the runTask isolation stuff, so it shooouuullldddd be ok. /** {@link #buildClassLoader(TaskToolbox)} has outdated javadocs referencing this field, TODO update */ @SuppressWarnings("unused") - protected static final Predicate IS_DRUID_URL = new Predicate() + protected static final Predicate IS_DRUID_URL = new Predicate<>() { @Override public boolean apply(@Nullable URL input) @@ -173,16 +173,7 @@ public static ClassLoader buildClassLoader(final List hadoopDependencyCo localClassLoaderURLs.addAll(Arrays.asList(hadoopLoader.getURLs())); } - ClassLoader parent = null; - if (JvmUtils.isIsJava9Compatible()) { - try { - // See also https://docs.oracle.com/en/java/javase/11/migrate/index.html#JSMIG-GUID-A868D0B9-026F-4D46-B979-901834343F9E - parent = (ClassLoader) ClassLoader.class.getMethod("getPlatformClassLoader").invoke(null); - } - catch (IllegalAccessException | InvocationTargetException | NoSuchMethodException e) { - throw new RuntimeException(e); - } - } + ClassLoader parent = ClassLoader.getPlatformClassLoader(); final ClassLoader classLoader = new URLClassLoader( localClassLoaderURLs.toArray(new URL[0]), parent diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/ParallelIndexPhaseRunner.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/ParallelIndexPhaseRunner.java index 96e4292926c2..579c1e492c38 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/ParallelIndexPhaseRunner.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/ParallelIndexPhaseRunner.java @@ -247,7 +247,7 @@ private void submitNewTask( final ListenableFuture> future = taskMonitor.submit(spec); Futures.addCallback( future, - new FutureCallback>() + new FutureCallback<>() { @Override public void onSuccess(SubTaskCompleteEvent completeEvent) diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/PartialDimensionCardinalityParallelIndexTaskRunner.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/PartialDimensionCardinalityParallelIndexTaskRunner.java index f297bfc2002f..686bf9e5dc8a 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/PartialDimensionCardinalityParallelIndexTaskRunner.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/PartialDimensionCardinalityParallelIndexTaskRunner.java @@ -68,7 +68,7 @@ SubTaskSpec createSubTaskSpec( ParallelIndexIngestionSpec subTaskIngestionSpec ) { - return new SubTaskSpec( + return new SubTaskSpec<>( id, groupId, supervisorTaskId, diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/PartialDimensionDistributionParallelIndexTaskRunner.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/PartialDimensionDistributionParallelIndexTaskRunner.java index a6c332b3907d..d18bdae97486 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/PartialDimensionDistributionParallelIndexTaskRunner.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/PartialDimensionDistributionParallelIndexTaskRunner.java @@ -295,7 +295,7 @@ SubTaskSpec createSubTaskSpec( ParallelIndexIngestionSpec subTaskIngestionSpec ) { - return new SubTaskSpec( + return new SubTaskSpec<>( id, groupId, supervisorTaskId, diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/PartialGenericSegmentMergeParallelIndexTaskRunner.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/PartialGenericSegmentMergeParallelIndexTaskRunner.java index 8babf50d8265..f469385ec546 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/PartialGenericSegmentMergeParallelIndexTaskRunner.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/PartialGenericSegmentMergeParallelIndexTaskRunner.java @@ -83,7 +83,7 @@ SubTaskSpec newTaskSpec(PartialSegmentMergeIOCon getTuningConfig() ); final String subtaskSpecId = getBaseSubtaskSpecName() + "_" + getAndIncrementNextSpecId(); - return new SubTaskSpec( + return new SubTaskSpec<>( subtaskSpecId, getGroupId(), getTaskId(), diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/PartialHashSegmentGenerateParallelIndexTaskRunner.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/PartialHashSegmentGenerateParallelIndexTaskRunner.java index 3bb9a1544f5d..dfd7e27466f2 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/PartialHashSegmentGenerateParallelIndexTaskRunner.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/PartialHashSegmentGenerateParallelIndexTaskRunner.java @@ -67,7 +67,7 @@ SubTaskSpec createSubTaskSpec( ParallelIndexIngestionSpec subTaskIngestionSpec ) { - return new SubTaskSpec( + return new SubTaskSpec<>( id, groupId, supervisorTaskId, diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/PartialRangeSegmentGenerateParallelIndexTaskRunner.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/PartialRangeSegmentGenerateParallelIndexTaskRunner.java index d37b432965b3..c6049e2379cb 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/PartialRangeSegmentGenerateParallelIndexTaskRunner.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/PartialRangeSegmentGenerateParallelIndexTaskRunner.java @@ -66,7 +66,7 @@ SubTaskSpec createSubTaskSpec( ParallelIndexIngestionSpec subTaskIngestionSpec ) { - return new SubTaskSpec( + return new SubTaskSpec<>( id, groupId, supervisorTaskId, diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/compact/LocalOverlordClient.java b/indexing-service/src/main/java/org/apache/druid/indexing/compact/LocalOverlordClient.java index dcce2f4615ae..3f1427c7c34e 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/compact/LocalOverlordClient.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/compact/LocalOverlordClient.java @@ -25,33 +25,25 @@ import com.google.common.util.concurrent.ListenableFuture; import org.apache.druid.client.indexing.ClientCompactionTaskQuery; import org.apache.druid.client.indexing.IndexingTotalWorkerCapacityInfo; -import org.apache.druid.client.indexing.IndexingWorkerInfo; import org.apache.druid.client.indexing.TaskPayloadResponse; -import org.apache.druid.client.indexing.TaskStatusResponse; import org.apache.druid.error.DruidException; import org.apache.druid.indexer.TaskStatus; import org.apache.druid.indexer.TaskStatusPlus; -import org.apache.druid.indexer.report.TaskReport; import org.apache.druid.indexing.common.task.CompactionTask; import org.apache.druid.indexing.overlord.TaskMaster; import org.apache.druid.indexing.overlord.TaskQueryTool; import org.apache.druid.indexing.overlord.TaskQueue; import org.apache.druid.indexing.overlord.http.TotalWorkerCapacityResponse; -import org.apache.druid.indexing.overlord.supervisor.SupervisorStatus; import org.apache.druid.java.util.common.CloseableIterators; import org.apache.druid.java.util.common.concurrent.Execs; import org.apache.druid.java.util.common.logger.Logger; import org.apache.druid.java.util.common.parsers.CloseableIterator; import org.apache.druid.metadata.LockFilterPolicy; -import org.apache.druid.rpc.ServiceRetryPolicy; -import org.apache.druid.rpc.indexing.OverlordClient; -import org.apache.druid.server.compaction.CompactionProgressResponse; -import org.apache.druid.server.compaction.CompactionStatusResponse; +import org.apache.druid.rpc.indexing.NoopOverlordClient; import org.joda.time.Interval; import javax.annotation.Nullable; import java.io.IOException; -import java.net.URI; import java.util.List; import java.util.Map; import java.util.Set; @@ -62,7 +54,7 @@ * task related info. This client simply redirects all queries to the * {@link TaskQueryTool} and all updates to the {@link TaskQueue}. */ -class LocalOverlordClient implements OverlordClient +class LocalOverlordClient extends NoopOverlordClient { private static final Logger log = new Logger(LocalOverlordClient.class); @@ -199,66 +191,4 @@ private V convertTask(Object taskPayload, Class inputType, Class ou ); } } - - // Unsupported methods as these are not used by the CompactionScheduler / CompactSegments duty - - @Override - public ListenableFuture findCurrentLeader() - { - throw new UnsupportedOperationException(); - } - - @Override - public ListenableFuture taskStatus(String taskId) - { - throw new UnsupportedOperationException(); - } - - @Override - public ListenableFuture taskReportAsMap(String taskId) - { - throw new UnsupportedOperationException(); - } - - @Override - public ListenableFuture> supervisorStatuses() - { - throw new UnsupportedOperationException(); - } - - @Override - public ListenableFuture killPendingSegments(String dataSource, Interval interval) - { - throw new UnsupportedOperationException(); - } - - @Override - public ListenableFuture> getWorkers() - { - throw new UnsupportedOperationException(); - } - - @Override - public ListenableFuture getCompactionSnapshots(@Nullable String dataSource) - { - throw new UnsupportedOperationException(); - } - - @Override - public ListenableFuture getBytesAwaitingCompaction(String dataSource) - { - throw new UnsupportedOperationException(); - } - - @Override - public ListenableFuture isCompactionSupervisorEnabled() - { - throw new UnsupportedOperationException(); - } - - @Override - public OverlordClient withRetryPolicy(ServiceRetryPolicy retryPolicy) - { - return this; - } } diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/input/DruidSegmentReader.java b/indexing-service/src/main/java/org/apache/druid/indexing/input/DruidSegmentReader.java index d2da00f38c87..2e4d64799aaa 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/input/DruidSegmentReader.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/input/DruidSegmentReader.java @@ -195,7 +195,7 @@ static CloseableIterator> makeCloseableIteratorFromSequenceA final CleanableFile segmentFile ) { - return new CloseableIterator>() + return new CloseableIterator<>() { Yielder> rowYielder = Yielders.each(sequence); diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/input/DruidTombstoneSegmentReader.java b/indexing-service/src/main/java/org/apache/druid/indexing/input/DruidTombstoneSegmentReader.java index a8b98dd7d0f8..39a0bc3848bd 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/input/DruidTombstoneSegmentReader.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/input/DruidTombstoneSegmentReader.java @@ -47,7 +47,7 @@ public DruidTombstoneSegmentReader( @Override protected CloseableIterator> intermediateRowIterator() { - return new CloseableIterator>() + return new CloseableIterator<>() { @Override public void close() diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/input/GeneratorInputSource.java b/indexing-service/src/main/java/org/apache/druid/indexing/input/GeneratorInputSource.java index b5dc706f3747..6228d341cf4c 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/input/GeneratorInputSource.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/input/GeneratorInputSource.java @@ -165,7 +165,7 @@ protected InputSourceReader fixedFormatReader(InputRowSchema inputRowSchema, @Nu @Override public CloseableIterator read(InputStats inputStats) { - return CloseableIterators.withEmptyBaggage(new Iterator() + return CloseableIterators.withEmptyBaggage(new Iterator<>() { int rowCount = 0; private final DataGenerator generator = makeGenerator(); @@ -191,7 +191,7 @@ public InputRow next() @Override public CloseableIterator sample() { - return CloseableIterators.withEmptyBaggage(new Iterator() + return CloseableIterators.withEmptyBaggage(new Iterator<>() { int rowCount = 0; private final DataGenerator generator = makeGenerator(); diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/overlord/DruidOverlord.java b/indexing-service/src/main/java/org/apache/druid/indexing/overlord/DruidOverlord.java index f8d011c70d80..547b63a70f07 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/overlord/DruidOverlord.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/overlord/DruidOverlord.java @@ -106,7 +106,7 @@ public void becomeLeader() giant.lock(); // I AM THE MASTER OF THE UNIVERSE. - log.info("By the power of Grayskull, I have the power!"); + log.info("By the power of Grayskull, I have the power. I am the leader"); try { final TaskRunner taskRunner = runnerFactory.build(); @@ -193,6 +193,7 @@ public void stopBeingLeader() giant.lock(); try { initialized = false; + log.info("I am no longer the leader..."); final Lifecycle leaderLifecycle = leaderLifecycleRef.getAndSet(null); if (leaderLifecycle != null) { diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/overlord/ForkingTaskRunner.java b/indexing-service/src/main/java/org/apache/druid/indexing/overlord/ForkingTaskRunner.java index 7e4dd6c39c22..d51d919a295e 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/overlord/ForkingTaskRunner.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/overlord/ForkingTaskRunner.java @@ -168,7 +168,7 @@ public ListenableFuture run(final Task task) new ForkingTaskRunnerWorkItem( task, exec.submit( - new Callable() { + new Callable<>() { @Override public TaskStatus call() { @@ -273,7 +273,7 @@ public TaskStatus call() try { List taskJavaOptsArray = jsonMapper.convertValue( task.getContextValue(ForkingTaskRunnerConfig.JAVA_OPTS_ARRAY_PROPERTY), - new TypeReference>() {} + new TypeReference<>() {} ); if (taskJavaOptsArray != null) { command.addAll(taskJavaOptsArray); diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/overlord/MetadataTaskStorage.java b/indexing-service/src/main/java/org/apache/druid/indexing/overlord/MetadataTaskStorage.java index 3608f0238992..683b9e805492 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/overlord/MetadataTaskStorage.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/overlord/MetadataTaskStorage.java @@ -57,30 +57,24 @@ public class MetadataTaskStorage implements TaskStorage { - private static final MetadataStorageActionHandlerTypes TASK_TYPES = new MetadataStorageActionHandlerTypes() + private static final MetadataStorageActionHandlerTypes TASK_TYPES = new MetadataStorageActionHandlerTypes<>() { @Override public TypeReference getEntryType() { - return new TypeReference() - { - }; + return new TypeReference<>() {}; } @Override public TypeReference getStatusType() { - return new TypeReference() - { - }; + return new TypeReference<>() {}; } @Override public TypeReference getLockType() { - return new TypeReference() - { - }; + return new TypeReference<>() {}; } }; diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/overlord/RemoteTaskRunner.java b/indexing-service/src/main/java/org/apache/druid/indexing/overlord/RemoteTaskRunner.java index 4fd7b57f38b5..4a4f0a084649 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/overlord/RemoteTaskRunner.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/overlord/RemoteTaskRunner.java @@ -248,7 +248,7 @@ public void start() } Futures.addCallback( addWorker(worker), - new FutureCallback() + new FutureCallback<>() { @Override public void onSuccess(ZkWorker zkWorker) diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/overlord/TaskLockbox.java b/indexing-service/src/main/java/org/apache/druid/indexing/overlord/TaskLockbox.java index 916f8cab75ca..1e1a43994925 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/overlord/TaskLockbox.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/overlord/TaskLockbox.java @@ -146,7 +146,7 @@ public TaskLockboxSyncResult syncFromStorage() } } // Sort locks by version, so we add them back in the order they were acquired. - final Ordering> byVersionOrdering = new Ordering>() + final Ordering> byVersionOrdering = new Ordering<>() { @Override public int compare(Pair left, Pair right) @@ -183,7 +183,7 @@ public int compare(Pair left, Pair right) ? savedTaskLock.withPriority(task.getPriority()) : savedTaskLock; - final TaskLockPosse taskLockPosse = verifyAndCreateOrFindLockPosse( + final TaskLockPosse taskLockPosse = reacquireLockOnStartup( task, savedTaskLockWithPriority ); @@ -192,15 +192,11 @@ public int compare(Pair left, Pair right) if (savedTaskLockWithPriority.getVersion().equals(taskLock.getVersion())) { taskLockCount++; - log.info( - "Reacquired lock[%s] for task: %s", - taskLock, - task.getId() - ); + log.info("Reacquired lock[%s] for task[%s].", taskLock, task.getId()); } else { taskLockCount++; log.info( - "Could not reacquire lock on interval[%s] version[%s] (got version[%s] instead) for task: %s", + "Could not reacquire lock on interval[%s] version[%s] (got version[%s] instead) for task[%s].", savedTaskLockWithPriority.getInterval(), savedTaskLockWithPriority.getVersion(), taskLock.getVersion(), @@ -210,7 +206,7 @@ public int compare(Pair left, Pair right) } else { failedToReacquireLockTaskGroups.add(task.getGroupId()); log.error( - "Could not reacquire lock on interval[%s] version[%s] for task: %s from group %s.", + "Could not reacquire lock on interval[%s] version[%s] for task[%s], groupId[%s].", savedTaskLockWithPriority.getInterval(), savedTaskLockWithPriority.getVersion(), task.getId(), @@ -253,38 +249,28 @@ public int compare(Pair left, Pair right) } /** - * This method is called only in {@link #syncFromStorage()} and verifies the given task and the taskLock have the same - * groupId, dataSource, and priority. + * Reacquire lock during {@link #syncFromStorage()}. + * + * @return null if the lock could not be reacquired. */ @VisibleForTesting @Nullable - protected TaskLockPosse verifyAndCreateOrFindLockPosse(Task task, TaskLock taskLock) + protected TaskLockPosse reacquireLockOnStartup(Task task, TaskLock taskLock) { + if (!taskMatchesLock(task, taskLock)) { + log.warn( + "Task[datasource: %s, groupId: %s, priority: %s] does not match" + + " TaskLock[datasource: %s, groupId: %s, priority: %s].", + task.getDataSource(), task.getGroupId(), task.getPriority(), + taskLock.getDataSource(), taskLock.getGroupId(), taskLock.getNonNullPriority() + ); + return null; + } + giant.lock(); try { - Preconditions.checkArgument( - task.getGroupId().equals(taskLock.getGroupId()), - "lock groupId[%s] is different from task groupId[%s]", - taskLock.getGroupId(), - task.getGroupId() - ); - Preconditions.checkArgument( - task.getDataSource().equals(taskLock.getDataSource()), - "lock dataSource[%s] is different from task dataSource[%s]", - taskLock.getDataSource(), - task.getDataSource() - ); final int taskPriority = task.getPriority(); - final int lockPriority = taskLock.getNonNullPriority(); - - Preconditions.checkArgument( - lockPriority == taskPriority, - "lock priority[%s] is different from task priority[%s]", - lockPriority, - taskPriority - ); - final LockRequest request; switch (taskLock.getGranularity()) { case SEGMENT: @@ -313,15 +299,13 @@ protected TaskLockPosse verifyAndCreateOrFindLockPosse(Task task, TaskLock taskL ); break; default: - throw new ISE("Unknown lockGranularity[%s]", taskLock.getGranularity()); + throw DruidException.defensive("Unknown lockGranularity[%s]", taskLock.getGranularity()); } return createOrFindLockPosse(request, task, false); } catch (Exception e) { - log.error(e, - "Could not reacquire lock for task: %s from metadata store", task.getId() - ); + log.error(e, "Could not reacquire lock for task[%s] from metadata store", task.getId()); return null; } finally { @@ -329,6 +313,17 @@ protected TaskLockPosse verifyAndCreateOrFindLockPosse(Task task, TaskLock taskL } } + /** + * Returns true if the datasource, groupId and priority of the given Task + * match that of the TaskLock. + */ + private boolean taskMatchesLock(Task task, TaskLock taskLock) + { + return task.getGroupId().equals(taskLock.getGroupId()) + && task.getDataSource().equals(taskLock.getDataSource()) + && task.getPriority() == taskLock.getNonNullPriority(); + } + /** * Acquires a lock on behalf of a task. Blocks until the lock is acquired. * @@ -751,13 +746,15 @@ private SegmentIdWithShardSpec allocateSegmentId(LockRequestForNewSegment reques { return metadataStorageCoordinator.allocatePendingSegment( request.getDataSource(), - request.getSequenceName(), - request.getPreviousSegmentId(), request.getInterval(), - request.getPartialShardSpec(), - version, request.isSkipSegmentLineageCheck(), - allocatorId + new SegmentCreateRequest( + request.getSequenceName(), + request.getPreviousSegmentId(), + version, + request.getPartialShardSpec(), + allocatorId + ) ); } @@ -1818,7 +1815,6 @@ SegmentCreateRequest getSegmentRequest() action.getPreviousSegmentId(), acquiredLock == null ? lockRequest.getVersion() : acquiredLock.getVersion(), action.getPartialShardSpec(), - null, ((PendingSegmentAllocatingTask) task).getTaskAllocatorId() ); } diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/overlord/TaskQueue.java b/indexing-service/src/main/java/org/apache/druid/indexing/overlord/TaskQueue.java index ea8f1a56ed85..94f691eecb75 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/overlord/TaskQueue.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/overlord/TaskQueue.java @@ -741,7 +741,7 @@ private void attachCallbacks(final Task task, final ListenableFuture Futures.addCallback( statusFuture, - new FutureCallback() + new FutureCallback<>() { @Override public void onSuccess(final TaskStatus status) diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/overlord/ThreadingTaskRunner.java b/indexing-service/src/main/java/org/apache/druid/indexing/overlord/ThreadingTaskRunner.java index 70b4a927af90..909163049f27 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/overlord/ThreadingTaskRunner.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/overlord/ThreadingTaskRunner.java @@ -151,7 +151,8 @@ public ListenableFuture run(Task task) new ThreadingTaskRunnerWorkItem( task, taskExecutor.submit( - new Callable() { + new Callable<>() + { @Override public TaskStatus call() { diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/overlord/autoscaling/PendingTaskBasedWorkerProvisioningStrategy.java b/indexing-service/src/main/java/org/apache/druid/indexing/overlord/autoscaling/PendingTaskBasedWorkerProvisioningStrategy.java index 6990973e564f..0d715b53549f 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/overlord/autoscaling/PendingTaskBasedWorkerProvisioningStrategy.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/overlord/autoscaling/PendingTaskBasedWorkerProvisioningStrategy.java @@ -117,7 +117,7 @@ public PendingTaskBasedWorkerProvisioningStrategy( config, workerConfigRef, provisioningSchedulerConfig, - new Supplier() + new Supplier<>() { @Override public ScheduledExecutorService get() @@ -178,7 +178,7 @@ public synchronized boolean doProvision() final Collection workerNodeIds = getWorkerNodeIDs( Collections2.transform( workers, - new Function() + new Function<>() { @Override public Worker apply(ImmutableWorkerInfo input) @@ -378,7 +378,7 @@ public synchronized boolean doTerminate() final Collection laziestWorkerIps = Collections2.transform( runner.markWorkersLazy(isLazyWorker, maxWorkersToTerminate), - new Function() + new Function<>() { @Override public String apply(Worker zkWorker) diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/overlord/autoscaling/ProvisioningUtil.java b/indexing-service/src/main/java/org/apache/druid/indexing/overlord/autoscaling/ProvisioningUtil.java index 6eb7e05305cf..162597e08b42 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/overlord/autoscaling/ProvisioningUtil.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/overlord/autoscaling/ProvisioningUtil.java @@ -29,7 +29,7 @@ public static Predicate createValidWorkerPredicate( final SimpleWorkerProvisioningConfig config ) { - return new Predicate() + return new Predicate<>() { @Override public boolean apply(ImmutableWorkerInfo worker) @@ -49,7 +49,7 @@ public static Predicate createLazyWorkerPredicate( { final Predicate isValidWorker = createValidWorkerPredicate(config); - return new Predicate() + return new Predicate<>() { @Override public boolean apply(ImmutableWorkerInfo worker) diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/overlord/autoscaling/SimpleWorkerProvisioningStrategy.java b/indexing-service/src/main/java/org/apache/druid/indexing/overlord/autoscaling/SimpleWorkerProvisioningStrategy.java index afdaa57c565d..fc0397393df7 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/overlord/autoscaling/SimpleWorkerProvisioningStrategy.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/overlord/autoscaling/SimpleWorkerProvisioningStrategy.java @@ -68,7 +68,7 @@ public SimpleWorkerProvisioningStrategy( config, workerConfigRef, provisioningSchedulerConfig, - new Supplier() + new Supplier<>() { @Override public ScheduledExecutorService get() @@ -133,7 +133,7 @@ public synchronized boolean doProvision() Lists.newArrayList( Iterables.transform( workers, - new Function() + new Function<>() { @Override public String apply(ImmutableWorkerInfo input) @@ -197,7 +197,7 @@ public synchronized boolean doTerminate() Lists.newArrayList( Iterables.transform( runner.getLazyWorkers(), - new Function() + new Function<>() { @Override public String apply(Worker input) @@ -223,7 +223,7 @@ public String apply(Worker input) final Collection laziestWorkerIps = Collections2.transform( runner.markWorkersLazy(isLazyWorker, excessWorkers), - new Function() + new Function<>() { @Override public String apply(Worker worker) diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/overlord/hrtr/WorkerHolder.java b/indexing-service/src/main/java/org/apache/druid/indexing/overlord/hrtr/WorkerHolder.java index 6294318ec54c..9d177c654109 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/overlord/hrtr/WorkerHolder.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/overlord/hrtr/WorkerHolder.java @@ -62,9 +62,7 @@ public class WorkerHolder { private static final EmittingLogger log = new EmittingLogger(WorkerHolder.class); - public static final TypeReference> WORKER_SYNC_RESP_TYPE_REF = new TypeReference>() - { - }; + public static final TypeReference> WORKER_SYNC_RESP_TYPE_REF = new TypeReference<>() {}; private final Worker worker; @@ -318,7 +316,7 @@ public ChangeRequestHttpSyncer getUnderlyingSyncer() public ChangeRequestHttpSyncer.Listener createSyncListener() { - return new ChangeRequestHttpSyncer.Listener() + return new ChangeRequestHttpSyncer.Listener<>() { @Override public void fullSync(List changes) diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/overlord/http/OverlordDataSourcesResource.java b/indexing-service/src/main/java/org/apache/druid/indexing/overlord/http/OverlordDataSourcesResource.java new file mode 100644 index 000000000000..bf9ff43981cd --- /dev/null +++ b/indexing-service/src/main/java/org/apache/druid/indexing/overlord/http/OverlordDataSourcesResource.java @@ -0,0 +1,285 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, + * software distributed under the License is distributed on an + * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY + * KIND, either express or implied. See the License for the + * specific language governing permissions and limitations + * under the License. + */ + +package org.apache.druid.indexing.overlord.http; + +import com.google.common.base.Throwables; +import com.google.common.collect.ImmutableMap; +import com.google.inject.Inject; +import com.sun.jersey.spi.container.ResourceFilters; +import org.apache.druid.audit.AuditEntry; +import org.apache.druid.audit.AuditManager; +import org.apache.druid.error.DruidException; +import org.apache.druid.error.InvalidInput; +import org.apache.druid.indexing.overlord.TaskMaster; +import org.apache.druid.java.util.common.StringUtils; +import org.apache.druid.java.util.common.logger.Logger; +import org.apache.druid.metadata.SegmentsMetadataManager; +import org.apache.druid.rpc.indexing.SegmentUpdateResponse; +import org.apache.druid.server.http.SegmentsToUpdateFilter; +import org.apache.druid.server.http.ServletResourceUtils; +import org.apache.druid.server.http.security.DatasourceResourceFilter; +import org.apache.druid.server.security.AuthorizationUtils; +import org.apache.druid.timeline.SegmentId; +import org.joda.time.Interval; + +import javax.servlet.http.HttpServletRequest; +import javax.ws.rs.Consumes; +import javax.ws.rs.DELETE; +import javax.ws.rs.POST; +import javax.ws.rs.Path; +import javax.ws.rs.PathParam; +import javax.ws.rs.Produces; +import javax.ws.rs.core.Context; +import javax.ws.rs.core.MediaType; +import javax.ws.rs.core.Response; +import java.util.ArrayList; +import java.util.List; +import java.util.Objects; +import java.util.Set; +import java.util.stream.Collectors; + +/** + * Datasource APIs exposed by the Overlord to update segments. + * Some of these APIs are also exposed by the Coordinator, but they have been + * deprecated and the Overlord APIs must be used for all update operations. + */ +@Path("/druid/indexer/v1/datasources") +public class OverlordDataSourcesResource +{ + private static final Logger log = new Logger(OverlordDataSourcesResource.class); + + private final SegmentsMetadataManager segmentsMetadataManager; + private final TaskMaster taskMaster; + private final AuditManager auditManager; + + @Inject + public OverlordDataSourcesResource( + TaskMaster taskMaster, + SegmentsMetadataManager segmentsMetadataManager, + AuditManager auditManager + ) + { + this.taskMaster = taskMaster; + this.auditManager = auditManager; + this.segmentsMetadataManager = segmentsMetadataManager; + } + + private interface SegmentUpdateOperation + { + int perform(); + } + + @POST + @Path("/{dataSourceName}") + @Consumes(MediaType.APPLICATION_JSON) + @ResourceFilters(DatasourceResourceFilter.class) + public Response markAllNonOvershadowedSegmentsAsUsed( + @PathParam("dataSourceName") final String dataSourceName, + @Context HttpServletRequest req + ) + { + SegmentUpdateOperation operation = () -> segmentsMetadataManager + .markAsUsedAllNonOvershadowedSegmentsInDataSource(dataSourceName); + return performSegmentUpdate(dataSourceName, operation); + } + + @DELETE + @Path("/{dataSourceName}") + @ResourceFilters(DatasourceResourceFilter.class) + @Produces(MediaType.APPLICATION_JSON) + public Response markAllSegmentsAsUnused( + @PathParam("dataSourceName") final String dataSourceName, + @Context HttpServletRequest req + ) + { + SegmentUpdateOperation operation = () -> segmentsMetadataManager + .markAsUnusedAllSegmentsInDataSource(dataSourceName); + final Response response = performSegmentUpdate(dataSourceName, operation); + + final int responseCode = response.getStatus(); + if (responseCode >= 200 && responseCode < 300) { + auditMarkUnusedOperation(response.getEntity(), dataSourceName, req); + } + + return response; + } + + @POST + @Path("/{dataSourceName}/markUsed") + @Consumes(MediaType.APPLICATION_JSON) + @ResourceFilters(DatasourceResourceFilter.class) + public Response markNonOvershadowedSegmentsAsUsed( + @PathParam("dataSourceName") final String dataSourceName, + final SegmentsToUpdateFilter payload + ) + { + if (payload == null || !payload.isValid()) { + return Response + .status(Response.Status.BAD_REQUEST) + .entity(SegmentsToUpdateFilter.INVALID_PAYLOAD_ERROR_MESSAGE) + .build(); + } else { + SegmentUpdateOperation operation = () -> { + final Interval interval = payload.getInterval(); + final List versions = payload.getVersions(); + if (interval != null) { + return segmentsMetadataManager.markAsUsedNonOvershadowedSegmentsInInterval(dataSourceName, interval, versions); + } else { + final Set segmentIds = payload.getSegmentIds(); + if (segmentIds == null || segmentIds.isEmpty()) { + return 0; + } + + // Validate segmentIds + final List invalidSegmentIds = new ArrayList<>(); + for (String segmentId : segmentIds) { + if (SegmentId.iteratePossibleParsingsWithDataSource(dataSourceName, segmentId).isEmpty()) { + invalidSegmentIds.add(segmentId); + } + } + if (!invalidSegmentIds.isEmpty()) { + throw InvalidInput.exception("Could not parse invalid segment IDs[%s]", invalidSegmentIds); + } + + return segmentsMetadataManager.markAsUsedNonOvershadowedSegments(dataSourceName, segmentIds); + } + }; + + return performSegmentUpdate(dataSourceName, operation); + } + } + + @POST + @Path("/{dataSourceName}/markUnused") + @ResourceFilters(DatasourceResourceFilter.class) + @Produces(MediaType.APPLICATION_JSON) + @Consumes(MediaType.APPLICATION_JSON) + public Response markSegmentsAsUnused( + @PathParam("dataSourceName") final String dataSourceName, + final SegmentsToUpdateFilter payload, + @Context final HttpServletRequest req + ) + { + if (payload == null || !payload.isValid()) { + return Response + .status(Response.Status.BAD_REQUEST) + .entity(SegmentsToUpdateFilter.INVALID_PAYLOAD_ERROR_MESSAGE) + .build(); + } else { + SegmentUpdateOperation operation = () -> { + final Interval interval = payload.getInterval(); + final List versions = payload.getVersions(); + final int numUpdatedSegments; + if (interval != null) { + numUpdatedSegments = segmentsMetadataManager.markAsUnusedSegmentsInInterval(dataSourceName, interval, versions); + } else { + final Set segmentIds = payload.getSegmentIds() + .stream() + .map(id -> SegmentId.tryParse(dataSourceName, id)) + .filter(Objects::nonNull) + .collect(Collectors.toSet()); + + // Filter out segmentIds that do not belong to this datasource + numUpdatedSegments = segmentsMetadataManager.markSegmentsAsUnused( + segmentIds.stream() + .filter(segmentId -> segmentId.getDataSource().equals(dataSourceName)) + .collect(Collectors.toSet()) + ); + } + auditMarkUnusedOperation(payload, dataSourceName, req); + return numUpdatedSegments; + }; + return performSegmentUpdate(dataSourceName, operation); + } + } + + @POST + @Path("/{dataSourceName}/segments/{segmentId}") + @Consumes(MediaType.APPLICATION_JSON) + @ResourceFilters(DatasourceResourceFilter.class) + public Response markSegmentAsUsed( + @PathParam("dataSourceName") String dataSourceName, + @PathParam("segmentId") String segmentId + ) + { + SegmentUpdateOperation operation = + () -> segmentsMetadataManager.markSegmentAsUsed(segmentId) ? 1 : 0; + return performSegmentUpdate(dataSourceName, operation); + } + + @DELETE + @Path("/{dataSourceName}/segments/{segmentId}") + @ResourceFilters(DatasourceResourceFilter.class) + public Response markSegmentAsUnused( + @PathParam("dataSourceName") String dataSourceName, + @PathParam("segmentId") String segmentIdString + ) + { + final SegmentId segmentId = SegmentId.tryParse(dataSourceName, segmentIdString); + if (segmentId == null) { + return Response.status(Response.Status.BAD_REQUEST).entity( + StringUtils.format("Could not parse Segment ID[%s] for DataSource[%s]", segmentIdString, dataSourceName) + ).build(); + } + + SegmentUpdateOperation operation = + () -> segmentsMetadataManager.markSegmentAsUnused(segmentId) ? 1 : 0; + return performSegmentUpdate(dataSourceName, operation); + } + + private Response performSegmentUpdate(String dataSourceName, SegmentUpdateOperation operation) + { + if (!taskMaster.isHalfOrFullLeader()) { + return Response.status(Response.Status.SERVICE_UNAVAILABLE).entity("I am not leader").build(); + } + + try { + int numChangedSegments = operation.perform(); + return Response.ok(new SegmentUpdateResponse(numChangedSegments)).build(); + } + catch (DruidException e) { + return ServletResourceUtils.buildErrorResponseFrom(e); + } + catch (Exception e) { + log.error(e, "Error occurred while updating segments for datasource[%s]", dataSourceName); + return Response + .serverError() + .entity(ImmutableMap.of("error", "Server error", "message", Throwables.getRootCause(e).toString())) + .build(); + } + } + + private void auditMarkUnusedOperation( + Object auditPayload, + String dataSourceName, + HttpServletRequest request + ) + { + auditManager.doAudit( + AuditEntry.builder() + .key(dataSourceName) + .type("segment.markUnused") + .payload(auditPayload) + .auditInfo(AuthorizationUtils.buildAuditInfo(request)) + .request(AuthorizationUtils.buildRequestInfo("overlord", request)) + .build() + ); + } +} diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/overlord/http/OverlordResource.java b/indexing-service/src/main/java/org/apache/druid/indexing/overlord/http/OverlordResource.java index 84bf260b6f99..fc2b00ad6f58 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/overlord/http/OverlordResource.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/overlord/http/OverlordResource.java @@ -512,7 +512,7 @@ public Response doAction(final TaskActionHolder holder) { return asLeaderWith( taskMaster.getTaskActionClient(holder.getTask()), - new Function() + new Function<>() { @Override public Response apply(TaskActionClient taskActionClient) diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/overlord/http/security/SupervisorResourceFilter.java b/indexing-service/src/main/java/org/apache/druid/indexing/overlord/http/security/SupervisorResourceFilter.java index 7e9aaa927407..c4be66719913 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/overlord/http/security/SupervisorResourceFilter.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/overlord/http/security/SupervisorResourceFilter.java @@ -64,7 +64,7 @@ public ContainerRequest filter(ContainerRequest request) .get( Iterables.indexOf( request.getPathSegments(), - new Predicate() + new Predicate<>() { @Override public boolean apply(PathSegment input) diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/overlord/supervisor/SupervisorManager.java b/indexing-service/src/main/java/org/apache/druid/indexing/overlord/supervisor/SupervisorManager.java index 243c41e4b5d8..731ddcaa1362 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/overlord/supervisor/SupervisorManager.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/overlord/supervisor/SupervisorManager.java @@ -21,7 +21,9 @@ import com.google.common.base.Optional; import com.google.common.base.Preconditions; +import com.google.common.util.concurrent.ListenableFuture; import com.google.inject.Inject; +import org.apache.druid.common.guava.FutureUtils; import org.apache.druid.error.DruidException; import org.apache.druid.indexing.common.TaskLockType; import org.apache.druid.indexing.common.task.Tasks; @@ -39,10 +41,12 @@ import org.apache.druid.segment.incremental.ParseExceptionReport; import javax.annotation.Nullable; +import java.util.ArrayList; import java.util.List; import java.util.Map; import java.util.Set; import java.util.concurrent.ConcurrentHashMap; +import java.util.concurrent.Future; /** * Manages the creation and lifetime of {@link Supervisor}. @@ -212,11 +216,12 @@ public void start() public void stop() { Preconditions.checkState(started, "SupervisorManager not started"); - + List> stopFutures = new ArrayList<>(); synchronized (lock) { + log.info("Stopping [%d] supervisors", supervisors.keySet().size()); for (String id : supervisors.keySet()) { try { - supervisors.get(id).lhs.stop(false); + stopFutures.add(supervisors.get(id).lhs.stopAsync()); SupervisorTaskAutoScaler autoscaler = autoscalers.get(id); if (autoscaler != null) { autoscaler.stop(); @@ -226,6 +231,18 @@ public void stop() log.warn(e, "Caught exception while stopping supervisor [%s]", id); } } + log.info("Waiting for [%d] supervisors to shutdown", stopFutures.size()); + try { + FutureUtils.coalesce(stopFutures).get(); + } + catch (Exception e) { + log.warn( + e, + "Stopped [%d] out of [%d] supervisors. Remaining supervisors will be killed.", + stopFutures.stream().filter(Future::isDone).count(), + stopFutures.size() + ); + } supervisors.clear(); autoscalers.clear(); started = false; diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/seekablestream/RecordSupplierInputSource.java b/indexing-service/src/main/java/org/apache/druid/indexing/seekablestream/RecordSupplierInputSource.java index b9bef3c7314f..96dda8ca88a7 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/seekablestream/RecordSupplierInputSource.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/seekablestream/RecordSupplierInputSource.java @@ -137,7 +137,7 @@ protected InputSourceReader formattableReader( */ CloseableIterator createEntityIterator() { - return new CloseableIterator() + return new CloseableIterator<>() { private Iterator> recordIterator; private Iterator bytesIterator; diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/seekablestream/SeekableStreamIndexTask.java b/indexing-service/src/main/java/org/apache/druid/indexing/seekablestream/SeekableStreamIndexTask.java index d43b83d78d07..7b10c0afdccf 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/seekablestream/SeekableStreamIndexTask.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/seekablestream/SeekableStreamIndexTask.java @@ -179,6 +179,16 @@ public QueryRunner getQueryRunner(Query query) return (queryPlus, responseContext) -> queryPlus.run(getRunner().getAppenderator(), responseContext); } + /** + * @return the current status of this task. + */ + @Nullable + public String getCurrentRunnerStatus() + { + SeekableStreamIndexTaskRunner.Status status = (getRunner() != null) ? getRunner().getStatus() : null; + return (status != null) ? status.toString() : null; + } + public Appenderator newAppenderator( TaskToolbox toolbox, SegmentGenerationMetrics metrics, diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/seekablestream/SeekableStreamIndexTaskClient.java b/indexing-service/src/main/java/org/apache/druid/indexing/seekablestream/SeekableStreamIndexTaskClient.java index 7fd282e44ce2..484ad15f7cd6 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/seekablestream/SeekableStreamIndexTaskClient.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/seekablestream/SeekableStreamIndexTaskClient.java @@ -32,7 +32,7 @@ public interface SeekableStreamIndexTaskClient { TypeReference> TYPE_REFERENCE_LIST_PARSE_EXCEPTION_REPORT = - new TypeReference>() {}; + new TypeReference<>() {}; /** * Retrieve current task checkpoints. diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/seekablestream/SeekableStreamIndexTaskClientAsyncImpl.java b/indexing-service/src/main/java/org/apache/druid/indexing/seekablestream/SeekableStreamIndexTaskClientAsyncImpl.java index 5de1cb50a971..6af2735e995b 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/seekablestream/SeekableStreamIndexTaskClientAsyncImpl.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/seekablestream/SeekableStreamIndexTaskClientAsyncImpl.java @@ -412,7 +412,7 @@ private ListenableFuture> getOffsetsWhe () -> Futures.addCallback( getOffsetsWhenPaused(taskId, retryPolicy), - new FutureCallback>() + new FutureCallback<>() { @Override public void onSuccess(@Nullable Map result) @@ -567,7 +567,7 @@ public ListenableFuture go() client.asyncRequest(requestBuilder.timeout(httpTimeout), responseHandler), responseTransformer ), - new FutureCallback() + new FutureCallback<>() { @Override public void onSuccess(@Nullable T result) diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/seekablestream/SeekableStreamIndexTaskClientFactory.java b/indexing-service/src/main/java/org/apache/druid/indexing/seekablestream/SeekableStreamIndexTaskClientFactory.java index c3b4e69f8327..92acb3a33296 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/seekablestream/SeekableStreamIndexTaskClientFactory.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/seekablestream/SeekableStreamIndexTaskClientFactory.java @@ -70,7 +70,7 @@ public SeekableStreamIndexTaskClient build( tuningConfig.getChatRetries() ); - return new SeekableStreamIndexTaskClientAsyncImpl( + return new SeekableStreamIndexTaskClientAsyncImpl<>( dataSource, new ServiceClientFactoryImpl(httpClient, connectExec), taskInfoProvider, diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/seekablestream/SeekableStreamIndexTaskRunner.java b/indexing-service/src/main/java/org/apache/druid/indexing/seekablestream/SeekableStreamIndexTaskRunner.java index a2db12d005e9..fddd8f8e4536 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/seekablestream/SeekableStreamIndexTaskRunner.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/seekablestream/SeekableStreamIndexTaskRunner.java @@ -711,7 +711,7 @@ public void run() if (isPersistRequired) { Futures.addCallback( driver.persistAsync(committerSupplier.get()), - new FutureCallback() + new FutureCallback<>() { @Override public void onSuccess(@Nullable Object result) @@ -1022,7 +1022,7 @@ private void publishAndRegisterHandoff(SequenceMetadata() + new FutureCallback<>() { @Override public void onSuccess(SegmentsAndCommitMetadata publishedSegmentsAndCommitMetadata) diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/seekablestream/SeekableStreamSamplerSpec.java b/indexing-service/src/main/java/org/apache/druid/indexing/seekablestream/SeekableStreamSamplerSpec.java index 4705f3337229..dddb46012891 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/seekablestream/SeekableStreamSamplerSpec.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/seekablestream/SeekableStreamSamplerSpec.java @@ -193,7 +193,7 @@ public SeekableStreamSamplerInputSourceReader(InputRowParser parser) @Override public CloseableIterator read() { - return new CloseableIterator() + return new CloseableIterator<>() { @Override @@ -225,7 +225,7 @@ public CloseableIterator read(InputStats inputStats) @Override public CloseableIterator sample() { - return new CloseableIterator() + return new CloseableIterator<>() { @Override public boolean hasNext() diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/seekablestream/supervisor/SeekableStreamSupervisor.java b/indexing-service/src/main/java/org/apache/druid/indexing/seekablestream/supervisor/SeekableStreamSupervisor.java index 8b4845b08d07..28e114fa2209 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/seekablestream/supervisor/SeekableStreamSupervisor.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/seekablestream/supervisor/SeekableStreamSupervisor.java @@ -34,6 +34,7 @@ import com.google.common.util.concurrent.AsyncFunction; import com.google.common.util.concurrent.Futures; import com.google.common.util.concurrent.ListenableFuture; +import com.google.common.util.concurrent.ListeningExecutorService; import com.google.common.util.concurrent.ListeningScheduledExecutorService; import com.google.common.util.concurrent.MoreExecutors; import it.unimi.dsi.fastutil.ints.Int2ObjectLinkedOpenHashMap; @@ -1103,6 +1104,19 @@ public void stop(boolean stopGracefully) } } + @Override + public ListenableFuture stopAsync() + { + ListeningExecutorService shutdownExec = MoreExecutors.listeningDecorator( + Execs.singleThreaded("supervisor-shutdown-" + StringUtils.encodeForFormat(supervisorId) + "--%d") + ); + return shutdownExec.submit(() -> { + stop(false); + shutdownExec.shutdown(); + return null; + }); + } + @Override public void reset(@Nullable final DataSourceMetadata dataSourceMetadata) { @@ -2064,7 +2078,7 @@ private void discoverTasks() throws ExecutionException, InterruptedException futures.add( Futures.transform( getStatusAndPossiblyEndOffsets(taskId), - new Function>, Boolean>() + new Function<>() { @Override public Boolean apply(Pair> pair) @@ -2100,7 +2114,8 @@ public Boolean apply(Pair stopTask(final String id, final boolean publish) { return Futures.transform( - taskClient.stopAsync(id, publish), new Function() + taskClient.stopAsync(id, publish), new Function<>() { @Nullable @Override @@ -3155,7 +3170,7 @@ private void updateTaskStatus() throws ExecutionException, InterruptedException futureTaskIds.add(taskId); futures.add( Futures.transform( - taskClient.getStartTimeAsync(taskId), new Function() + taskClient.getStartTimeAsync(taskId), new Function<>() { @Override public Boolean apply(@Nullable DateTime startTime) @@ -3377,7 +3392,7 @@ public Map apply(@Nullable Object input) return Futures.transformAsync( FutureUtils.coalesce(pauseFutures), - new AsyncFunction>>, Map>() + new AsyncFunction<>() { @Override public ListenableFuture> apply(List>> input) diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/worker/WorkerTaskManager.java b/indexing-service/src/main/java/org/apache/druid/indexing/worker/WorkerTaskManager.java index a1c131ad0f49..8b1153376791 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/worker/WorkerTaskManager.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/worker/WorkerTaskManager.java @@ -245,7 +245,7 @@ private void addRunningTask(final Task task, final ListenableFuture runningTasks.put(task.getId(), new TaskDetails(task)); Futures.addCallback( future, - new FutureCallback() + new FutureCallback<>() { @Override public void onSuccess(TaskStatus result) diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/worker/executor/ExecutorLifecycle.java b/indexing-service/src/main/java/org/apache/druid/indexing/worker/executor/ExecutorLifecycle.java index 8784ee044618..44a2d5508ca5 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/worker/executor/ExecutorLifecycle.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/worker/executor/ExecutorLifecycle.java @@ -176,7 +176,7 @@ public void start() throws InterruptedException statusFuture = Futures.transform( taskRunner.run(task), - new Function() + new Function<>() { @Override public TaskStatus apply(TaskStatus taskStatus) diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/common/TaskToolboxTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/common/TaskToolboxTest.java index 78cbb88c3cf4..25aa3c4a0899 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/common/TaskToolboxTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/common/TaskToolboxTest.java @@ -24,7 +24,6 @@ import org.apache.druid.client.cache.CacheConfig; import org.apache.druid.client.cache.CachePopulatorStats; import org.apache.druid.client.coordinator.NoopCoordinatorClient; -import org.apache.druid.client.indexing.NoopOverlordClient; import org.apache.druid.indexing.common.actions.TaskActionClientFactory; import org.apache.druid.indexing.common.config.TaskConfig; import org.apache.druid.indexing.common.config.TaskConfigBuilder; @@ -39,6 +38,7 @@ import org.apache.druid.query.DruidProcessingConfigTest; import org.apache.druid.query.QueryProcessingPool; import org.apache.druid.query.QueryRunnerFactoryConglomerate; +import org.apache.druid.rpc.indexing.NoopOverlordClient; import org.apache.druid.segment.IndexIO; import org.apache.druid.segment.IndexMergerV9; import org.apache.druid.segment.IndexMergerV9Factory; diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/common/TestUtils.java b/indexing-service/src/test/java/org/apache/druid/indexing/common/TestUtils.java index ae5ba44f7f09..ea58e21d035f 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/common/TestUtils.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/common/TestUtils.java @@ -25,19 +25,18 @@ import com.fasterxml.jackson.databind.module.SimpleModule; import com.google.common.base.Stopwatch; import com.google.common.collect.ImmutableMap; -import org.apache.druid.client.indexing.NoopOverlordClient; import org.apache.druid.data.input.impl.NoopInputFormat; import org.apache.druid.data.input.impl.NoopInputSource; import org.apache.druid.guice.DruidSecondaryModule; import org.apache.druid.indexing.common.stats.DropwizardRowIngestionMetersFactory; import org.apache.druid.indexing.common.task.TestAppenderatorsManager; -import org.apache.druid.indexing.common.task.batch.parallel.ParallelIndexSupervisorTaskClientProvider; import org.apache.druid.jackson.DefaultObjectMapper; import org.apache.druid.java.util.common.ISE; import org.apache.druid.java.util.common.StringUtils; import org.apache.druid.java.util.common.logger.Logger; import org.apache.druid.math.expr.ExprMacroTable; import org.apache.druid.query.expression.LookupEnabledTestExprMacroTable; +import org.apache.druid.rpc.indexing.NoopOverlordClient; import org.apache.druid.rpc.indexing.OverlordClient; import org.apache.druid.segment.IndexIO; import org.apache.druid.segment.IndexMergerV9; @@ -63,10 +62,6 @@ public class TestUtils { public static final OverlordClient OVERLORD_SERVICE_CLIENT = new NoopOverlordClient(); - public static final ParallelIndexSupervisorTaskClientProvider TASK_CLIENT_PROVIDER = - (supervisorTaskId, httpTimeout, numRetries) -> { - throw new UnsupportedOperationException(); - }; public static final AppenderatorsManager APPENDERATORS_MANAGER = new TestAppenderatorsManager(); private static final Logger log = new Logger(TestUtils.class); diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/common/actions/SegmentTransactionalInsertActionTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/common/actions/SegmentTransactionalInsertActionTest.java index 44ce60b5ceb2..7d5f4488fea0 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/common/actions/SegmentTransactionalInsertActionTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/common/actions/SegmentTransactionalInsertActionTest.java @@ -22,7 +22,6 @@ import com.google.common.collect.ImmutableList; import com.google.common.collect.ImmutableMap; import com.google.common.collect.ImmutableSet; -import org.apache.druid.error.InvalidInput; import org.apache.druid.indexing.common.TaskLockType; import org.apache.druid.indexing.common.task.NoopTask; import org.apache.druid.indexing.common.task.Task; @@ -32,6 +31,7 @@ import org.apache.druid.indexing.overlord.Segments; import org.apache.druid.indexing.overlord.TimeChunkLockRequest; import org.apache.druid.java.util.common.Intervals; +import org.apache.druid.metadata.RetryTransactionException; import org.apache.druid.timeline.DataSegment; import org.apache.druid.timeline.partition.LinearShardSpec; import org.assertj.core.api.Assertions; @@ -151,7 +151,7 @@ public void testFailTransactionalUpdateDataSourceMetadata() throws Exception Assert.assertEquals( SegmentPublishResult.fail( - InvalidInput.exception( + new RetryTransactionException( "The new start metadata state[ObjectMetadata{theObject=[1]}] is" + " ahead of the last committed end state[null]. Try resetting the supervisor." ).toString() diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/ClientCompactionTaskQuerySerdeTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/ClientCompactionTaskQuerySerdeTest.java index 55898357d663..a354cb20db28 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/ClientCompactionTaskQuerySerdeTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/ClientCompactionTaskQuerySerdeTest.java @@ -35,7 +35,6 @@ import org.apache.druid.client.indexing.ClientCompactionTaskQueryTuningConfig; import org.apache.druid.client.indexing.ClientCompactionTaskTransformSpec; import org.apache.druid.client.indexing.ClientTaskQuery; -import org.apache.druid.client.indexing.NoopOverlordClient; import org.apache.druid.data.input.SegmentsSplitHintSpec; import org.apache.druid.data.input.impl.DimensionsSpec; import org.apache.druid.guice.GuiceAnnotationIntrospector; @@ -53,6 +52,7 @@ import org.apache.druid.query.aggregation.AggregatorFactory; import org.apache.druid.query.aggregation.CountAggregatorFactory; import org.apache.druid.query.filter.SelectorDimFilter; +import org.apache.druid.rpc.indexing.NoopOverlordClient; import org.apache.druid.rpc.indexing.OverlordClient; import org.apache.druid.segment.IndexSpec; import org.apache.druid.segment.TestIndex; diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/CompactionTaskRunTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/CompactionTaskRunTest.java index a3fb807604fa..e6e9230439cc 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/CompactionTaskRunTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/CompactionTaskRunTest.java @@ -32,7 +32,6 @@ import org.apache.druid.client.coordinator.NoopCoordinatorClient; import org.apache.druid.client.indexing.ClientCompactionTaskGranularitySpec; import org.apache.druid.client.indexing.ClientCompactionTaskTransformSpec; -import org.apache.druid.client.indexing.NoopOverlordClient; import org.apache.druid.data.input.impl.CSVParseSpec; import org.apache.druid.data.input.impl.DimensionSchema; import org.apache.druid.data.input.impl.DimensionsSpec; @@ -70,6 +69,7 @@ import org.apache.druid.query.aggregation.LongSumAggregatorFactory; import org.apache.druid.query.dimension.DefaultDimensionSpec; import org.apache.druid.query.filter.SelectorDimFilter; +import org.apache.druid.rpc.indexing.NoopOverlordClient; import org.apache.druid.rpc.indexing.OverlordClient; import org.apache.druid.segment.AutoTypeColumnSchema; import org.apache.druid.segment.ColumnSelectorFactory; diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/FilteringCloseableInputRowIteratorTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/FilteringCloseableInputRowIteratorTest.java index a0b1c101d40c..2b4ffe26f31b 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/FilteringCloseableInputRowIteratorTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/FilteringCloseableInputRowIteratorTest.java @@ -95,7 +95,7 @@ public void testFilterOutRows() public void testParseExceptionInDelegateNext() { // This iterator throws ParseException every other call to next(). - final CloseableIterator parseExceptionThrowingIterator = new CloseableIterator() + final CloseableIterator parseExceptionThrowingIterator = new CloseableIterator<>() { final int numRowsToIterate = ROWS.size() * 2; int nextIdx = 0; @@ -144,7 +144,7 @@ public void testParseExceptionInPredicateTest() ROWS.iterator() ); // This filter throws ParseException every other call to test(). - final Predicate filter = new Predicate() + final Predicate filter = new Predicate<>() { boolean throwParseException = false; @@ -183,7 +183,7 @@ public boolean test(InputRow inputRow) public void testParseExceptionInDelegateHasNext() { // This iterator throws ParseException every other call to hasNext(). - final CloseableIterator parseExceptionThrowingIterator = new CloseableIterator() + final CloseableIterator parseExceptionThrowingIterator = new CloseableIterator<>() { final int numRowsToIterate = ROWS.size() * 2; int currentIndex = 0; @@ -229,7 +229,7 @@ public void close() public void testNonParseExceptionInDelegateHasNext() { // This iterator throws ParseException every other call to hasNext(). - final CloseableIterator parseExceptionThrowingIterator = new CloseableIterator() + final CloseableIterator parseExceptionThrowingIterator = new CloseableIterator<>() { final int numRowsToIterate = ROWS.size() * 2; int currentIndex = 0; @@ -294,7 +294,7 @@ public void testParseExceptionSaveExceptionCause() { // This iterator throws ParseException every other call to hasNext(). - final CloseableIterator parseExceptionThrowingIterator = new CloseableIterator() + final CloseableIterator parseExceptionThrowingIterator = new CloseableIterator<>() { final int numRowsToIterate = ROWS.size() * 2; int currentIndex = 0; diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/HadoopTaskTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/HadoopTaskTest.java index 130258e85d4c..741208dd0db4 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/HadoopTaskTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/HadoopTaskTest.java @@ -30,7 +30,6 @@ import org.apache.druid.java.util.common.StringUtils; import org.apache.druid.java.util.common.granularity.Granularity; import org.apache.druid.timeline.DataSegment; -import org.apache.druid.utils.JvmUtils; import org.easymock.EasyMock; import org.joda.time.Interval; import org.junit.Assert; @@ -131,13 +130,8 @@ public TaskStatus runTask(TaskToolbox toolbox) public static void assertClassLoaderIsSingular(ClassLoader classLoader) { - if (JvmUtils.isIsJava9Compatible()) { - // See also https://docs.oracle.com/en/java/javase/11/migrate/index.html#JSMIG-GUID-A868D0B9-026F-4D46-B979-901834343F9E - Assert.assertEquals("PlatformClassLoader", classLoader.getParent().getClass().getSimpleName()); - } else { - // This is a check against the current HadoopTask which creates a single URLClassLoader with null parent - Assert.assertNull(classLoader.getParent()); - } + // See also https://docs.oracle.com/en/java/javase/11/migrate/index.html#JSMIG-GUID-A868D0B9-026F-4D46-B979-901834343F9E + Assert.assertEquals("PlatformClassLoader", classLoader.getParent().getClass().getSimpleName()); Assert.assertFalse(classLoader.getClass().getSimpleName().equals("ApplicationClassLoader")); Assert.assertTrue(classLoader instanceof URLClassLoader); diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/IndexTaskTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/IndexTaskTest.java index 7fbd6317d82f..d1e54665f2d7 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/IndexTaskTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/IndexTaskTest.java @@ -2551,9 +2551,7 @@ private IngestionStatsAndErrors getTaskReportData() throws IOException { TaskReport.ReportMap taskReports = jsonMapper.readValue( taskRunner.getTaskReportsFile(), - new TypeReference() - { - } + new TypeReference<>() {} ); return IngestionStatsAndErrors.getPayloadFromTaskReports(taskReports); } diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/AbstractParallelIndexSupervisorTaskTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/AbstractParallelIndexSupervisorTaskTest.java index d96a4d2a37e9..9dd6fe3f88a9 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/AbstractParallelIndexSupervisorTaskTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/AbstractParallelIndexSupervisorTaskTest.java @@ -33,7 +33,6 @@ import org.apache.druid.client.ImmutableDruidDataSource; import org.apache.druid.client.coordinator.CoordinatorClient; import org.apache.druid.client.coordinator.NoopCoordinatorClient; -import org.apache.druid.client.indexing.NoopOverlordClient; import org.apache.druid.client.indexing.TaskStatusResponse; import org.apache.druid.data.input.InputFormat; import org.apache.druid.data.input.MaxSizeSplitHintSpec; @@ -83,6 +82,7 @@ import org.apache.druid.query.aggregation.AggregatorFactory; import org.apache.druid.query.aggregation.LongSumAggregatorFactory; import org.apache.druid.query.expression.LookupEnabledTestExprMacroTable; +import org.apache.druid.rpc.indexing.NoopOverlordClient; import org.apache.druid.segment.DataSegmentsWithSchemas; import org.apache.druid.segment.IndexIO; import org.apache.druid.segment.TestIndex; diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/ParallelIndexPhaseRunnerTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/ParallelIndexPhaseRunnerTest.java index ade1692ca1c4..5e0b09103c76 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/ParallelIndexPhaseRunnerTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/ParallelIndexPhaseRunnerTest.java @@ -142,7 +142,7 @@ private static class TestPhaseRunner extends ParallelIndexPhaseRunner> subTaskSpecIterator() { - return new Iterator>() + return new Iterator<>() { int subTaskCount = 0; diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/TaskMonitorTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/TaskMonitorTest.java index 4d04dbd86cd8..fdbb7bb0a526 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/TaskMonitorTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/TaskMonitorTest.java @@ -21,7 +21,6 @@ import com.google.common.util.concurrent.Futures; import com.google.common.util.concurrent.ListenableFuture; -import org.apache.druid.client.indexing.NoopOverlordClient; import org.apache.druid.client.indexing.TaskStatusResponse; import org.apache.druid.data.input.InputSplit; import org.apache.druid.indexer.RunnerTaskState; @@ -35,6 +34,7 @@ import org.apache.druid.java.util.common.DateTimes; import org.apache.druid.java.util.common.ISE; import org.apache.druid.java.util.common.concurrent.Execs; +import org.apache.druid.rpc.indexing.NoopOverlordClient; import org.junit.After; import org.junit.Assert; import org.junit.Before; diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/iterator/IndexTaskInputRowIteratorBuilderTestingFactory.java b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/iterator/IndexTaskInputRowIteratorBuilderTestingFactory.java index a1821318ba56..dab24fbb3a03 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/iterator/IndexTaskInputRowIteratorBuilderTestingFactory.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/iterator/IndexTaskInputRowIteratorBuilderTestingFactory.java @@ -63,7 +63,7 @@ static InputRow createInputRow(DateTime timestamp, List dimensionValues) static CloseableIterator createInputRowIterator(InputRow inputRow) { - return new CloseableIterator() + return new CloseableIterator<>() { @Override public void close() diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/input/DruidSegmentReaderTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/input/DruidSegmentReaderTest.java index 5dcc11c9f1c5..0875cdde8677 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/input/DruidSegmentReaderTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/input/DruidSegmentReaderTest.java @@ -646,7 +646,7 @@ public void testMakeCloseableIteratorFromSequenceAndSegmentFileCloseYielderOnClo MutableBoolean isSequenceClosed = new MutableBoolean(false); MutableBoolean isFileClosed = new MutableBoolean(false); Sequence> sequence = new BaseSequence<>( - new IteratorMaker, Iterator>>() + new IteratorMaker<>() { @Override public Iterator> make() diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/overlord/OverlordBlinkLeadershipTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/overlord/OverlordBlinkLeadershipTest.java index 3824cc20f85e..af3b81977998 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/overlord/OverlordBlinkLeadershipTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/overlord/OverlordBlinkLeadershipTest.java @@ -36,7 +36,7 @@ public class OverlordBlinkLeadershipTest private RemoteTaskRunnerTestUtils rtrUtils; private final TestRemoteTaskRunnerConfig remoteTaskRunnerConfig = new TestRemoteTaskRunnerConfig(new Period("PT5M")); private final DefaultWorkerBehaviorConfig defaultWorkerBehaviourConfig = DefaultWorkerBehaviorConfig.defaultConfig(); - private final Supplier workerBehaviorConfigSupplier = new Supplier() + private final Supplier workerBehaviorConfigSupplier = new Supplier<>() { @Override public DefaultWorkerBehaviorConfig get() diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/overlord/RemoteTaskRunnerTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/overlord/RemoteTaskRunnerTest.java index 2ab63cd79ee5..8207584f2136 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/overlord/RemoteTaskRunnerTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/overlord/RemoteTaskRunnerTest.java @@ -454,7 +454,7 @@ public void testBootstrap() throws Exception final Set runningTasks = Sets.newHashSet( Iterables.transform( remoteTaskRunner.getRunningTasks(), - new Function() + new Function<>() { @Override public String apply(RemoteTaskRunnerWorkItem input) @@ -721,7 +721,7 @@ public void testFindLazyWorkerTaskRunning() throws Exception Assert.assertTrue(taskAnnounced(task.getId())); mockWorkerRunningTask(task); Collection lazyworkers = remoteTaskRunner.markWorkersLazy( - new Predicate() + new Predicate<>() { @Override public boolean apply(ImmutableWorkerInfo input) @@ -742,7 +742,7 @@ public void testFindLazyWorkerForWorkerJustAssignedTask() throws Exception remoteTaskRunner.run(task); Assert.assertTrue(taskAnnounced(task.getId())); Collection lazyworkers = remoteTaskRunner.markWorkersLazy( - new Predicate() + new Predicate<>() { @Override public boolean apply(ImmutableWorkerInfo input) @@ -761,7 +761,7 @@ public void testFindLazyWorkerNotRunningAnyTask() throws Exception { doSetup(); Collection lazyworkers = remoteTaskRunner.markWorkersLazy( - new Predicate() + new Predicate<>() { @Override public boolean apply(ImmutableWorkerInfo input) @@ -782,7 +782,7 @@ public void testFindLazyWorkerNotRunningAnyTaskButWithZeroMaxWorkers() throws Ex { doSetup(); Collection lazyworkers = remoteTaskRunner.markWorkersLazy( - new Predicate() + new Predicate<>() { @Override public boolean apply(ImmutableWorkerInfo input) diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/overlord/SingleTaskBackgroundRunnerTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/overlord/SingleTaskBackgroundRunnerTest.java index 3acf9ff6f0e5..5590ce608ec8 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/overlord/SingleTaskBackgroundRunnerTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/overlord/SingleTaskBackgroundRunnerTest.java @@ -21,7 +21,6 @@ import com.google.common.util.concurrent.ListenableFuture; import org.apache.druid.client.coordinator.NoopCoordinatorClient; -import org.apache.druid.client.indexing.NoopOverlordClient; import org.apache.druid.indexer.TaskLocation; import org.apache.druid.indexer.TaskState; import org.apache.druid.indexer.TaskStatus; @@ -45,6 +44,7 @@ import org.apache.druid.query.QueryRunner; import org.apache.druid.query.scan.ScanResultValue; import org.apache.druid.query.spec.MultipleIntervalSegmentSpec; +import org.apache.druid.rpc.indexing.NoopOverlordClient; import org.apache.druid.segment.TestIndex; import org.apache.druid.segment.join.NoopJoinableFactory; import org.apache.druid.segment.loading.NoopDataSegmentArchiver; diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/overlord/TaskLifecycleTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/overlord/TaskLifecycleTest.java index abab38bf0c9b..8400e27d2359 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/overlord/TaskLifecycleTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/overlord/TaskLifecycleTest.java @@ -35,7 +35,6 @@ import org.apache.druid.client.cache.CachePopulatorStats; import org.apache.druid.client.cache.MapCache; import org.apache.druid.client.coordinator.NoopCoordinatorClient; -import org.apache.druid.client.indexing.NoopOverlordClient; import org.apache.druid.data.input.AbstractInputSource; import org.apache.druid.data.input.InputRow; import org.apache.druid.data.input.InputRowListPlusRawValues; @@ -110,6 +109,7 @@ import org.apache.druid.query.QueryRunnerFactoryConglomerate; import org.apache.druid.query.SegmentDescriptor; import org.apache.druid.query.aggregation.DoubleSumAggregatorFactory; +import org.apache.druid.rpc.indexing.NoopOverlordClient; import org.apache.druid.segment.IndexIO; import org.apache.druid.segment.IndexMergerV9Factory; import org.apache.druid.segment.IndexSpec; @@ -201,7 +201,7 @@ public TaskLifecycleTest(String taskStorageType) @Rule public final TemporaryFolder temporaryFolder = new TemporaryFolder(); - private static final Ordering BY_INTERVAL_ORDERING = new Ordering() + private static final Ordering BY_INTERVAL_ORDERING = new Ordering<>() { @Override public int compare(DataSegment dataSegment, DataSegment dataSegment2) @@ -287,7 +287,7 @@ protected InputSourceReader fixedFormatReader(InputRowSchema inputRowSchema, @Nu @Override public CloseableIterator read(InputStats inputStats) { - return new CloseableIterator() + return new CloseableIterator<>() { @Override public void close() @@ -777,7 +777,7 @@ public void testKillUnusedSegmentsTask() throws Exception "2011-04-01/2011-04-02", "2011-04-02/2011-04-03", "2011-04-04/2011-04-05" - ), new Function() + ), new Function<>() { @Override public DataSegment apply(String input) @@ -874,7 +874,7 @@ public void testKillUnusedSegmentsTaskWithMaxSegmentsToKill() throws Exception "2011-04-01/2011-04-02", "2011-04-02/2011-04-03", "2011-04-04/2011-04-05" - ), new Function() + ), new Function<>() { @Override public DataSegment apply(String input) diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/overlord/TaskLockboxTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/overlord/TaskLockboxTest.java index a8c4b5117b1b..8f47b78a3bfe 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/overlord/TaskLockboxTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/overlord/TaskLockboxTest.java @@ -2270,10 +2270,10 @@ public NullLockPosseTaskLockbox( } @Override - protected TaskLockPosse verifyAndCreateOrFindLockPosse(Task task, TaskLock taskLock) + protected TaskLockPosse reacquireLockOnStartup(Task task, TaskLock taskLock) { return task.getGroupId() - .contains("FailingLockAcquisition") ? null : super.verifyAndCreateOrFindLockPosse(task, taskLock); + .contains("FailingLockAcquisition") ? null : super.reacquireLockOnStartup(task, taskLock); } } } diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/overlord/TestTaskRunner.java b/indexing-service/src/test/java/org/apache/druid/indexing/overlord/TestTaskRunner.java index 2b97b170be36..2afb2b4dae31 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/overlord/TestTaskRunner.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/overlord/TestTaskRunner.java @@ -219,7 +219,7 @@ public ListenableFuture run(final Task task) runningItems.add(taskRunnerWorkItem); Futures.addCallback( statusFuture, - new FutureCallback() + new FutureCallback<>() { @Override public void onSuccess(TaskStatus result) diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/overlord/TestTaskToolboxFactory.java b/indexing-service/src/test/java/org/apache/druid/indexing/overlord/TestTaskToolboxFactory.java index 19e4bd33fd8e..3f3081b0e69e 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/overlord/TestTaskToolboxFactory.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/overlord/TestTaskToolboxFactory.java @@ -26,7 +26,6 @@ import org.apache.druid.client.cache.CachePopulatorStats; import org.apache.druid.client.coordinator.CoordinatorClient; import org.apache.druid.client.coordinator.NoopCoordinatorClient; -import org.apache.druid.client.indexing.NoopOverlordClient; import org.apache.druid.discovery.DataNodeService; import org.apache.druid.discovery.DruidNodeAnnouncer; import org.apache.druid.discovery.LookupNodeService; @@ -44,6 +43,7 @@ import org.apache.druid.java.util.metrics.MonitorScheduler; import org.apache.druid.query.QueryProcessingPool; import org.apache.druid.query.QueryRunnerFactoryConglomerate; +import org.apache.druid.rpc.indexing.NoopOverlordClient; import org.apache.druid.rpc.indexing.OverlordClient; import org.apache.druid.segment.IndexIO; import org.apache.druid.segment.IndexMergerV9Factory; diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/overlord/autoscaling/PendingTaskBasedProvisioningStrategyTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/overlord/autoscaling/PendingTaskBasedProvisioningStrategyTest.java index 249f510d818c..8a1232b95714 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/overlord/autoscaling/PendingTaskBasedProvisioningStrategyTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/overlord/autoscaling/PendingTaskBasedProvisioningStrategyTest.java @@ -101,7 +101,7 @@ public void setUp() config, DSuppliers.of(workerConfig), new ProvisioningSchedulerConfig(), - new Supplier() + new Supplier<>() { @Override public ScheduledExecutorService get() @@ -127,7 +127,7 @@ public void testGetExpectedWorkerCapacityWithNoWorkerAndHintIsValid() config, DSuppliers.of(workerConfig), new ProvisioningSchedulerConfig(), - new Supplier() + new Supplier<>() { @Override public ScheduledExecutorService get() @@ -155,7 +155,7 @@ public void testGetExpectedWorkerCapacityWithNoWorkerAndHintIsNotValid() config, DSuppliers.of(workerConfig), new ProvisioningSchedulerConfig(), - new Supplier() + new Supplier<>() { @Override public ScheduledExecutorService get() @@ -278,7 +278,7 @@ public void testProvisionNoCurrentlyRunningWorkerWithCapacityHintSetAndNoPending config, DSuppliers.of(workerConfig), new ProvisioningSchedulerConfig(), - new Supplier() + new Supplier<>() { @Override public ScheduledExecutorService get() @@ -329,7 +329,7 @@ public void testProvisionNoCurrentlyRunningWorkerWithCapacityHintSetAndNoPending config, DSuppliers.of(workerConfig), new ProvisioningSchedulerConfig(), - new Supplier() + new Supplier<>() { @Override public ScheduledExecutorService get() @@ -493,7 +493,7 @@ public void testProvisionWithPendingTaskAndWorkerCapacityHintSetButNonEmptyCurre config, DSuppliers.of(workerConfig), new ProvisioningSchedulerConfig(), - new Supplier() + new Supplier<>() { @Override public ScheduledExecutorService get() @@ -568,7 +568,7 @@ public void testProvisionWithPendingTaskAndWorkerCapacityHintSetButEmptyCurrentl config, DSuppliers.of(workerConfig), new ProvisioningSchedulerConfig(), - new Supplier() + new Supplier<>() { @Override public ScheduledExecutorService get() diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/overlord/autoscaling/SimpleProvisioningStrategyTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/overlord/autoscaling/SimpleProvisioningStrategyTest.java index 7f9739ad55ff..0799b3ba6e43 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/overlord/autoscaling/SimpleProvisioningStrategyTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/overlord/autoscaling/SimpleProvisioningStrategyTest.java @@ -93,7 +93,7 @@ public void setUp() simpleWorkerProvisioningConfig, DSuppliers.of(workerConfig), schedulerConfig, - new Supplier() + new Supplier<>() { @Override public ScheduledExecutorService get() diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/overlord/http/OverlordDataSourcesResourceTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/overlord/http/OverlordDataSourcesResourceTest.java new file mode 100644 index 000000000000..7bea4b2aec1b --- /dev/null +++ b/indexing-service/src/test/java/org/apache/druid/indexing/overlord/http/OverlordDataSourcesResourceTest.java @@ -0,0 +1,370 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, + * software distributed under the License is distributed on an + * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY + * KIND, either express or implied. See the License for the + * specific language governing permissions and limitations + * under the License. + */ + +package org.apache.druid.indexing.overlord.http; + +import com.google.common.collect.ImmutableSet; +import org.apache.druid.audit.AuditManager; +import org.apache.druid.client.ImmutableDruidDataSource; +import org.apache.druid.indexing.overlord.TaskMaster; +import org.apache.druid.java.util.common.Intervals; +import org.apache.druid.java.util.common.granularity.Granularities; +import org.apache.druid.rpc.indexing.SegmentUpdateResponse; +import org.apache.druid.segment.TestDataSource; +import org.apache.druid.server.coordinator.CreateDataSegments; +import org.apache.druid.server.coordinator.simulate.TestSegmentsMetadataManager; +import org.apache.druid.server.http.SegmentsToUpdateFilter; +import org.apache.druid.server.security.AuthConfig; +import org.apache.druid.timeline.DataSegment; +import org.easymock.EasyMock; +import org.joda.time.Interval; +import org.junit.Assert; +import org.junit.Before; +import org.junit.Test; + +import javax.servlet.http.HttpServletRequest; +import javax.ws.rs.core.Response; +import java.util.Collection; +import java.util.Collections; +import java.util.List; +import java.util.Set; +import java.util.stream.Collectors; + +public class OverlordDataSourcesResourceTest +{ + private static final String WIKI_SEGMENTS_START = "2024-01-01"; + private static final List WIKI_SEGMENTS_10X1D + = CreateDataSegments.ofDatasource(TestDataSource.WIKI) + .forIntervals(10, Granularities.DAY) + .startingAt(WIKI_SEGMENTS_START) + .eachOfSizeInMb(500); + + private TestSegmentsMetadataManager segmentsMetadataManager; + + private OverlordDataSourcesResource dataSourcesResource; + + @Before + public void setup() + { + AuditManager auditManager = EasyMock.createStrictMock(AuditManager.class); + segmentsMetadataManager = new TestSegmentsMetadataManager(); + + TaskMaster taskMaster = new TaskMaster(null, null); + dataSourcesResource = new OverlordDataSourcesResource( + taskMaster, + segmentsMetadataManager, + auditManager + ); + taskMaster.becomeFullLeader(); + + WIKI_SEGMENTS_10X1D.forEach(segmentsMetadataManager::addSegment); + } + + @Test + public void testMarkSegmentAsUnused() + { + Response response = dataSourcesResource.markSegmentAsUnused( + TestDataSource.WIKI, + WIKI_SEGMENTS_10X1D.get(0).getId().toString() + ); + verifyNumSegmentsUpdated(1, response); + } + + @Test + public void testMarkSegmentAsUnused_withInvalidSegmentId() + { + Response response = dataSourcesResource.markSegmentAsUnused( + TestDataSource.WIKI, + "someSegment" + ); + Assert.assertEquals(400, response.getStatus()); + Assert.assertEquals( + "Could not parse Segment ID[someSegment] for DataSource[wiki]", + response.getEntity() + ); + } + + @Test + public void testMarkAllSegmentsAsUnused() + { + Response response = dataSourcesResource.markAllSegmentsAsUnused( + TestDataSource.WIKI, + createHttpServletRequest() + ); + verifyNumSegmentsUpdated(10, response); + } + + @Test + public void testMarkSegmentsAsUnused_bySegmentIds() + { + final Set segmentIdsToUpdate = ImmutableSet.of( + WIKI_SEGMENTS_10X1D.get(0).getId().toString(), + WIKI_SEGMENTS_10X1D.get(8).getId().toString() + ); + + Response response = dataSourcesResource.markSegmentsAsUnused( + TestDataSource.WIKI, + new SegmentsToUpdateFilter(null, segmentIdsToUpdate, null), + createHttpServletRequest() + ); + verifyNumSegmentsUpdated(2, response); + } + + @Test + public void testMarkSegmentsAsUnused_byInterval() + { + final Interval nonOverlappingInterval = Intervals.of("1000/2000"); + Response response = dataSourcesResource.markSegmentsAsUnused( + TestDataSource.WIKI, + new SegmentsToUpdateFilter(nonOverlappingInterval, null, null), + createHttpServletRequest() + ); + verifyNumSegmentsUpdated(0, response); + + final Interval overlappingInterval3Days = Intervals.of(WIKI_SEGMENTS_START + "/P3D"); + response = dataSourcesResource.markSegmentsAsUnused( + TestDataSource.WIKI, + new SegmentsToUpdateFilter(overlappingInterval3Days, null, null), + createHttpServletRequest() + ); + verifyNumSegmentsUpdated(3, response); + } + + @Test + public void testMarkSegmentsAsUnused_byIntervalAndVersion() + { + final Interval overlappingInterval3Days = Intervals.of(WIKI_SEGMENTS_START + "/P3D"); + Response response = dataSourcesResource.markSegmentsAsUnused( + TestDataSource.WIKI, + new SegmentsToUpdateFilter( + overlappingInterval3Days, + null, + Collections.singletonList("invalidVersion") + ), + createHttpServletRequest() + ); + verifyNumSegmentsUpdated(0, response); + + final String wikiSegmentVersion = WIKI_SEGMENTS_10X1D.get(0).getVersion(); + response = dataSourcesResource.markSegmentsAsUnused( + TestDataSource.WIKI, + new SegmentsToUpdateFilter( + overlappingInterval3Days, + null, + Collections.singletonList(wikiSegmentVersion) + ), + createHttpServletRequest() + ); + verifyNumSegmentsUpdated(3, response); + } + + @Test + public void testMarkSegmentAsUsed() + { + final DataSegment segment = WIKI_SEGMENTS_10X1D.get(0); + final String segmentId = segment.getId().toString(); + + // Verify that segment which is already "used" is not updated + Response response = dataSourcesResource.markSegmentAsUsed(TestDataSource.WIKI, segmentId); + verifyNumSegmentsUpdated(0, response); + + // Mark segment as unused and then mark it as used + dataSourcesResource.markSegmentAsUnused(TestDataSource.WIKI, segmentId); + response = dataSourcesResource.markSegmentAsUsed(TestDataSource.WIKI, segmentId); + verifyNumSegmentsUpdated(1, response); + } + + @Test + public void testMarkAllNonOvershadowedSegmentsAsUsed() + { + // Create higher version segments + final String version2 = WIKI_SEGMENTS_10X1D.get(0).getVersion() + "__2"; + final List wikiSegmentsV2 = WIKI_SEGMENTS_10X1D.stream().map( + segment -> DataSegment.builder(segment).version(version2).build() + ).collect(Collectors.toList()); + + wikiSegmentsV2.forEach(segmentsMetadataManager::addSegment); + + // Mark all segments as unused + Response response = dataSourcesResource.markAllSegmentsAsUnused( + TestDataSource.WIKI, + createHttpServletRequest() + ); + verifyNumSegmentsUpdated(20, response); + + // Verify that only higher version segments are marked as used + response = dataSourcesResource.markAllNonOvershadowedSegmentsAsUsed( + TestDataSource.WIKI, + createHttpServletRequest() + ); + verifyNumSegmentsUpdated(10, response); + + final ImmutableDruidDataSource dataSource = segmentsMetadataManager + .getImmutableDataSourceWithUsedSegments(TestDataSource.WIKI); + Assert.assertNotNull(dataSource); + + final Collection usedSegments = dataSource.getSegments(); + Assert.assertEquals(10, usedSegments.size()); + for (DataSegment segment : usedSegments) { + Assert.assertEquals(version2, segment.getVersion()); + } + } + + @Test + public void testMarkNonOvershadowedSegmentsAsUsed_byInterval() + { + dataSourcesResource.markAllSegmentsAsUnused(TestDataSource.WIKI, createHttpServletRequest()); + + final Interval disjointInterval = Intervals.of("1000/2000"); + Response response = dataSourcesResource.markNonOvershadowedSegmentsAsUsed( + TestDataSource.WIKI, + new SegmentsToUpdateFilter(disjointInterval, null, null) + ); + verifyNumSegmentsUpdated(0, response); + + final Interval overlappingInterval3Days = Intervals.of(WIKI_SEGMENTS_START + "/P3D"); + response = dataSourcesResource.markNonOvershadowedSegmentsAsUsed( + TestDataSource.WIKI, + new SegmentsToUpdateFilter(overlappingInterval3Days, null, null) + ); + verifyNumSegmentsUpdated(3, response); + } + + @Test + public void testMarkNonOvershadowedSegmentsAsUsed_byIntervalAndVersion() + { + dataSourcesResource.markAllSegmentsAsUnused(TestDataSource.WIKI, createHttpServletRequest()); + + final Interval overlappingInterval4Days = Intervals.of(WIKI_SEGMENTS_START + "/P4D"); + Response response = dataSourcesResource.markNonOvershadowedSegmentsAsUsed( + TestDataSource.WIKI, + new SegmentsToUpdateFilter( + overlappingInterval4Days, + null, + Collections.singletonList("invalidVersion") + ) + ); + verifyNumSegmentsUpdated(0, response); + + final String wikiSegmentsVersion = WIKI_SEGMENTS_10X1D.get(0).getVersion(); + response = dataSourcesResource.markNonOvershadowedSegmentsAsUsed( + TestDataSource.WIKI, + new SegmentsToUpdateFilter( + overlappingInterval4Days, + null, + Collections.singletonList(wikiSegmentsVersion) + ) + ); + verifyNumSegmentsUpdated(4, response); + } + + @Test + public void testMarkNonOvershadowedSegmentsAsUsed_bySegmentIds() + { + dataSourcesResource.markAllSegmentsAsUnused(TestDataSource.WIKI, createHttpServletRequest()); + + final Set segmentIdsToUpdate = ImmutableSet.of( + WIKI_SEGMENTS_10X1D.get(0).getId().toString(), + WIKI_SEGMENTS_10X1D.get(1).getId().toString() + ); + Response response = dataSourcesResource.markNonOvershadowedSegmentsAsUsed( + TestDataSource.WIKI, + new SegmentsToUpdateFilter(null, segmentIdsToUpdate, null) + ); + verifyNumSegmentsUpdated(2, response); + } + + @Test + public void testMarkNonOvershadowedSegmentsAsUsed_withNullPayload_throwsBadRequestError() + { + final Response response + = dataSourcesResource.markNonOvershadowedSegmentsAsUsed(TestDataSource.WIKI, null); + Assert.assertEquals(400, response.getStatus()); + Assert.assertEquals( + "Invalid request payload. Specify either 'interval' or 'segmentIds', but not both." + + " Optionally, include 'versions' only when 'interval' is provided.", + response.getEntity() + ); + } + + @Test + public void testMarkNonOvershadowedSegmentsAsUsed_withInvalidPayload_throwsBadRequestError() + { + final String segmentId = WIKI_SEGMENTS_10X1D.get(0).getId().toString(); + final String expectedErrorMessage + = "Invalid request payload. Specify either 'interval' or 'segmentIds', but not both." + + " Optionally, include 'versions' only when 'interval' is provided."; + + // Both interval and segmentIds are null + Response response = dataSourcesResource.markNonOvershadowedSegmentsAsUsed( + TestDataSource.WIKI, + new SegmentsToUpdateFilter(null, null, null) + ); + Assert.assertEquals(400, response.getStatus()); + Assert.assertEquals(expectedErrorMessage, response.getEntity()); + + // interval is null and segmentIds is empty + response = dataSourcesResource.markNonOvershadowedSegmentsAsUsed( + TestDataSource.WIKI, + new SegmentsToUpdateFilter(null, Collections.emptySet(), null) + ); + Assert.assertEquals(400, response.getStatus()); + Assert.assertEquals(expectedErrorMessage, response.getEntity()); + + // Both interval and segmentIds are specified + response = dataSourcesResource.markNonOvershadowedSegmentsAsUsed( + TestDataSource.WIKI, + new SegmentsToUpdateFilter(Intervals.of("1000/2000"), Collections.singleton(segmentId), null) + ); + Assert.assertEquals(400, response.getStatus()); + Assert.assertEquals(expectedErrorMessage, response.getEntity()); + + // versions are specified with segmentIds + response = dataSourcesResource.markNonOvershadowedSegmentsAsUsed( + TestDataSource.WIKI, + new SegmentsToUpdateFilter(null, Collections.singleton(segmentId), Collections.singletonList("v1")) + ); + Assert.assertEquals(400, response.getStatus()); + Assert.assertEquals(expectedErrorMessage, response.getEntity()); + } + + private void verifyNumSegmentsUpdated(int expectedUpdatedCount, Response response) + { + Assert.assertEquals(200, response.getStatus()); + Assert.assertEquals(new SegmentUpdateResponse(expectedUpdatedCount), response.getEntity()); + } + + private static HttpServletRequest createHttpServletRequest() + { + final HttpServletRequest request = EasyMock.createStrictMock(HttpServletRequest.class); + + EasyMock.expect(request.getHeader(AuditManager.X_DRUID_AUTHOR)).andReturn("author").anyTimes(); + EasyMock.expect(request.getHeader(AuditManager.X_DRUID_COMMENT)).andReturn("comment").anyTimes(); + EasyMock.expect(request.getAttribute(AuthConfig.DRUID_AUTHENTICATION_RESULT)).andReturn(null).anyTimes(); + EasyMock.expect(request.getRemoteAddr()).andReturn("127.0.0.1").anyTimes(); + + EasyMock.expect(request.getMethod()).andReturn("POST").anyTimes(); + EasyMock.expect(request.getRequestURI()).andReturn("/request/uri").anyTimes(); + EasyMock.expect(request.getQueryString()).andReturn("query=string").anyTimes(); + + EasyMock.replay(request); + + return request; + } +} diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/overlord/http/OverlordResourceTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/overlord/http/OverlordResourceTest.java index 732d586002f8..2837f63e0590 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/overlord/http/OverlordResourceTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/overlord/http/OverlordResourceTest.java @@ -1076,9 +1076,7 @@ public void testGetLockedIntervals() throws Exception final ObjectMapper jsonMapper = TestHelper.makeJsonMapper(); Map> observedIntervals = jsonMapper.readValue( jsonMapper.writeValueAsString(response.getEntity()), - new TypeReference>>() - { - } + new TypeReference<>() {} ); Assert.assertEquals(expectedIntervals, observedIntervals); diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/overlord/http/OverlordTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/overlord/http/OverlordTest.java index f1a8b65a509f..8c5e9e924cc0 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/overlord/http/OverlordTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/overlord/http/OverlordTest.java @@ -215,7 +215,7 @@ public void setUp() throws Exception taskCompletionCountDownLatches.put(badTaskId, new CountDownLatch(1)); taskCompletionCountDownLatches.put(goodTaskId, new CountDownLatch(1)); - TaskRunnerFactory taskRunnerFactory = new TaskRunnerFactory() + TaskRunnerFactory taskRunnerFactory = new TaskRunnerFactory<>() { private MockTaskRunner runner; @@ -440,7 +440,7 @@ public synchronized ListenableFuture run(final Task task) "noop_test_task_exec_%s" ) ).submit( - new Callable() + new Callable<>() { @Override public TaskStatus call() throws Exception @@ -499,7 +499,7 @@ public synchronized Collection getRunningTasks() { return Lists.transform( runningTasks, - new Function() + new Function<>() { @Nullable @Override diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/overlord/sampler/InputSourceSamplerDiscoveryTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/overlord/sampler/InputSourceSamplerDiscoveryTest.java index 0220aacd8922..463416fc33e6 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/overlord/sampler/InputSourceSamplerDiscoveryTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/overlord/sampler/InputSourceSamplerDiscoveryTest.java @@ -31,7 +31,6 @@ import org.apache.druid.data.input.impl.StringDimensionSchema; import org.apache.druid.data.input.impl.TimestampSpec; import org.apache.druid.jackson.DefaultObjectMapper; -import org.apache.druid.math.expr.ExpressionProcessing; import org.apache.druid.segment.AutoTypeColumnSchema; import org.apache.druid.segment.column.ColumnType; import org.apache.druid.segment.column.RowSignature; @@ -56,71 +55,6 @@ public class InputSourceSamplerDiscoveryTest extends InitializedNullHandlingTest ); private InputSourceSampler inputSourceSampler = new InputSourceSampler(OBJECT_MAPPER); - @Test - public void testDiscoveredTypesNonStrictBooleans() - { - - try { - ExpressionProcessing.initializeForStrictBooleansTests(false); - final InputSource inputSource = new InlineInputSource(Strings.join(STR_JSON_ROWS, '\n')); - final SamplerResponse response = inputSourceSampler.sample( - inputSource, - new JsonInputFormat(null, null, null, null, null), - DataSchema.builder() - .withDataSource("test") - .withTimestamp(new TimestampSpec("t", null, null)) - .withDimensions(DimensionsSpec.builder().useSchemaDiscovery(true).build()) - .build(), - null - ); - - Assert.assertEquals(6, response.getNumRowsRead()); - Assert.assertEquals(5, response.getNumRowsIndexed()); - Assert.assertEquals(6, response.getData().size()); - Assert.assertEquals( - ImmutableList.of( - new StringDimensionSchema("string"), - new LongDimensionSchema("long"), - new DoubleDimensionSchema("double"), - new StringDimensionSchema("bool"), - new StringDimensionSchema("variant"), - new AutoTypeColumnSchema("array", null), - new AutoTypeColumnSchema("nested", null) - ), - response.getLogicalDimensions() - ); - - Assert.assertEquals( - ImmutableList.of( - new AutoTypeColumnSchema("string", null), - new AutoTypeColumnSchema("long", null), - new AutoTypeColumnSchema("double", null), - new AutoTypeColumnSchema("bool", null), - new AutoTypeColumnSchema("variant", null), - new AutoTypeColumnSchema("array", null), - new AutoTypeColumnSchema("nested", null) - ), - response.getPhysicalDimensions() - ); - Assert.assertEquals( - RowSignature.builder() - .addTimeColumn() - .add("string", ColumnType.STRING) - .add("long", ColumnType.LONG) - .add("double", ColumnType.DOUBLE) - .add("bool", ColumnType.STRING) - .add("variant", ColumnType.STRING) - .add("array", ColumnType.LONG_ARRAY) - .add("nested", ColumnType.NESTED_DATA) - .build(), - response.getLogicalSegmentSchema() - ); - } - finally { - ExpressionProcessing.initializeForTests(); - } - } - @Test public void testDiscoveredTypesStrictBooleans() { diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/overlord/supervisor/SupervisorManagerTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/overlord/supervisor/SupervisorManagerTest.java index 81153f238a4e..add6d4473dec 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/overlord/supervisor/SupervisorManagerTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/overlord/supervisor/SupervisorManagerTest.java @@ -23,6 +23,7 @@ import com.google.common.collect.ImmutableList; import com.google.common.collect.ImmutableMap; import com.google.common.collect.ImmutableSet; +import com.google.common.util.concurrent.SettableFuture; import org.apache.druid.error.DruidException; import org.apache.druid.error.DruidExceptionMatcher; import org.apache.druid.indexing.common.TaskLockType; @@ -130,7 +131,9 @@ public void testCreateUpdateAndRemoveSupervisor() verifyAll(); resetAll(); - supervisor3.stop(false); + SettableFuture stopFuture = SettableFuture.create(); + stopFuture.set(null); + EasyMock.expect(supervisor3.stopAsync()).andReturn(stopFuture); replayAll(); manager.stop(); @@ -361,7 +364,7 @@ public void testStopThrowsException() EasyMock.expect(metadataSupervisorManager.getLatest()).andReturn(existingSpecs); supervisor1.start(); - supervisor1.stop(false); + supervisor1.stopAsync(); EasyMock.expectLastCall().andThrow(new RuntimeException("RTE")); replayAll(); @@ -511,7 +514,9 @@ public void testCreateSuspendResumeAndStopSupervisor() // mock manager shutdown to ensure supervisor 3 stops resetAll(); - supervisor3.stop(false); + SettableFuture stopFuture = SettableFuture.create(); + stopFuture.set(null); + EasyMock.expect(supervisor3.stopAsync()).andReturn(stopFuture); replayAll(); manager.stop(); diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/seekablestream/SeekableStreamEndSequenceNumbersTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/seekablestream/SeekableStreamEndSequenceNumbersTest.java index 9cf29d6cbd25..6f0f5d3b51eb 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/seekablestream/SeekableStreamEndSequenceNumbersTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/seekablestream/SeekableStreamEndSequenceNumbersTest.java @@ -50,7 +50,7 @@ public void testSerde() throws Exception // Check round-trip. final SeekableStreamEndSequenceNumbers partitions2 = OBJECT_MAPPER.readValue( serializedString, - new TypeReference>() {} + new TypeReference<>() {} ); Assert.assertEquals("Round trip", partitions, partitions2); diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/seekablestream/SeekableStreamIndexTaskTestBase.java b/indexing-service/src/test/java/org/apache/druid/indexing/seekablestream/SeekableStreamIndexTaskTestBase.java index 7a4fd7dadb3a..b815a48d71e6 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/seekablestream/SeekableStreamIndexTaskTestBase.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/seekablestream/SeekableStreamIndexTaskTestBase.java @@ -34,7 +34,6 @@ import org.apache.druid.client.cache.CachePopulatorStats; import org.apache.druid.client.cache.MapCache; import org.apache.druid.client.coordinator.NoopCoordinatorClient; -import org.apache.druid.client.indexing.NoopOverlordClient; import org.apache.druid.common.config.NullHandling; import org.apache.druid.data.input.InputFormat; import org.apache.druid.data.input.impl.ByteEntity; @@ -100,6 +99,7 @@ import org.apache.druid.query.aggregation.LongSumAggregatorFactory; import org.apache.druid.query.timeseries.TimeseriesQuery; import org.apache.druid.query.timeseries.TimeseriesResultValue; +import org.apache.druid.rpc.indexing.NoopOverlordClient; import org.apache.druid.segment.DimensionHandlerUtils; import org.apache.druid.segment.IndexIO; import org.apache.druid.segment.QueryableIndex; diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/seekablestream/SeekableStreamStartSequenceNumbersTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/seekablestream/SeekableStreamStartSequenceNumbersTest.java index f632cf61ecf2..9f93648e3821 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/seekablestream/SeekableStreamStartSequenceNumbersTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/seekablestream/SeekableStreamStartSequenceNumbersTest.java @@ -51,7 +51,7 @@ public void testSerde() throws Exception // Check round-trip. final SeekableStreamStartSequenceNumbers partitions2 = OBJECT_MAPPER.readValue( serializedString, - new TypeReference>() {} + new TypeReference<>() {} ); Assert.assertEquals("Round trip", partitions, partitions2); diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/seekablestream/SeekableStreamSupervisorSpecTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/seekablestream/SeekableStreamSupervisorSpecTest.java index 5f5aeb0ca853..425b38045970 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/seekablestream/SeekableStreamSupervisorSpecTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/seekablestream/SeekableStreamSupervisorSpecTest.java @@ -188,7 +188,7 @@ protected SeekableStreamIndexTaskIOConfig createTaskIoConfig( SeekableStreamSupervisorIOConfig ioConfig ) { - return new SeekableStreamIndexTaskIOConfig( + return new SeekableStreamIndexTaskIOConfig<>( groupId, baseSequenceName, new SeekableStreamStartSequenceNumbers<>(STREAM, startPartitions, exclusiveStartSequenceNumberPartitions), @@ -247,7 +247,7 @@ protected SeekableStreamDataSourceMetadata createDataSourceMetaD @Override protected OrderedSequenceNumber makeSequenceNumber(String seq, boolean isExclusive) { - return new OrderedSequenceNumber(seq, isExclusive) + return new OrderedSequenceNumber<>(seq, isExclusive) { @Override public int compareTo(OrderedSequenceNumber o) @@ -281,7 +281,7 @@ protected SeekableStreamSupervisorReportPayload createReportPayl boolean includeOffsets ) { - return new SeekableStreamSupervisorReportPayload( + return new SeekableStreamSupervisorReportPayload<>( DATASOURCE, STREAM, 1, diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/seekablestream/SequenceMetadataTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/seekablestream/SequenceMetadataTest.java index cae9ec1e686f..d65a21ddf942 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/seekablestream/SequenceMetadataTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/seekablestream/SequenceMetadataTest.java @@ -164,7 +164,7 @@ public void testCanHandle() private OrderedSequenceNumber makeSequenceNumber(String seq, boolean isExclusive) { - return new OrderedSequenceNumber(seq, isExclusive) + return new OrderedSequenceNumber<>(seq, isExclusive) { @Override public int compareTo(OrderedSequenceNumber o) diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/seekablestream/supervisor/SeekableStreamSupervisorStateManagerTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/seekablestream/supervisor/SeekableStreamSupervisorStateManagerTest.java index 4d4eaccb60c5..6ffe9dcdfc5d 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/seekablestream/supervisor/SeekableStreamSupervisorStateManagerTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/seekablestream/supervisor/SeekableStreamSupervisorStateManagerTest.java @@ -372,9 +372,7 @@ public void testExceptionEventSerde() throws IOException String serialized = defaultMapper.writeValueAsString(event); - Map deserialized = defaultMapper.readValue(serialized, new TypeReference>() - { - }); + Map deserialized = defaultMapper.readValue(serialized, new TypeReference<>() {}); Assert.assertNotNull(deserialized.get("timestamp")); Assert.assertEquals("java.lang.NullPointerException", deserialized.get("exceptionClass")); Assert.assertFalse(Boolean.getBoolean(deserialized.get("streamException"))); diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/seekablestream/supervisor/SeekableStreamSupervisorStateTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/seekablestream/supervisor/SeekableStreamSupervisorStateTest.java index 8a3d36ea67fe..b1600d3ad549 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/seekablestream/supervisor/SeekableStreamSupervisorStateTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/seekablestream/supervisor/SeekableStreamSupervisorStateTest.java @@ -143,6 +143,7 @@ public class SeekableStreamSupervisorStateTest extends EasyMockSupport private SeekableStreamSupervisorSpec spec; private SeekableStreamIndexTaskClient indexTaskClient; private RecordSupplier recordSupplier; + private SeekableStreamIndexTaskRunner streamingTaskRunner; private RowIngestionMetersFactory rowIngestionMetersFactory; private SupervisorStateManagerConfig supervisorConfig; @@ -161,6 +162,7 @@ public void setupTest() spec = createMock(SeekableStreamSupervisorSpec.class); indexTaskClient = createMock(SeekableStreamIndexTaskClient.class); recordSupplier = createMock(RecordSupplier.class); + streamingTaskRunner = createMock(SeekableStreamIndexTaskRunner.class); rowIngestionMetersFactory = new TestUtils().getRowIngestionMetersFactory(); @@ -1000,6 +1002,28 @@ public void testStopping() verifyAll(); } + @Test + public void testStopGracefully() throws Exception + { + EasyMock.expect(spec.isSuspended()).andReturn(false).anyTimes(); + EasyMock.expect(recordSupplier.getPartitionIds(STREAM)).andReturn(ImmutableSet.of(SHARD_ID)).anyTimes(); + EasyMock.expect(taskStorage.getActiveTasksByDatasource(DATASOURCE)).andReturn(ImmutableList.of()).anyTimes(); + EasyMock.expect(taskQueue.add(EasyMock.anyObject())).andReturn(true).anyTimes(); + + taskRunner.unregisterListener("testSupervisorId"); + indexTaskClient.close(); + recordSupplier.close(); + + replayAll(); + SeekableStreamSupervisor supervisor = new TestSeekableStreamSupervisor(); + + supervisor.start(); + supervisor.runInternal(); + ListenableFuture stopFuture = supervisor.stopAsync(); + stopFuture.get(); + verifyAll(); + } + @Test public void testStoppingGracefully() { @@ -1459,6 +1483,8 @@ public Duration getEmissionDuration() supervisor.start(); supervisor.runInternal(); + Assert.assertNull(id1.getCurrentRunnerStatus()); + supervisor.checkpoint( 0, new TestSeekableStreamDataSourceMetadata( @@ -1518,6 +1544,8 @@ public Duration getEmissionDuration() EasyMock.expect(spec.getContextValue("tags")).andReturn("").anyTimes(); EasyMock.expect(recordSupplier.getPartitionIds(STREAM)).andReturn(ImmutableSet.of(SHARD_ID)).anyTimes(); EasyMock.expect(taskQueue.add(EasyMock.anyObject())).andReturn(true).anyTimes(); + EasyMock.expect(streamingTaskRunner.getStatus()).andReturn(null); + EasyMock.expect(streamingTaskRunner.getStatus()).andReturn(SeekableStreamIndexTaskRunner.Status.NOT_STARTED); SeekableStreamIndexTaskTuningConfig taskTuningConfig = getTuningConfig().convertToTaskTuningConfig(); @@ -1543,7 +1571,8 @@ public Duration getEmissionDuration() ioConfig ), context, - "0" + "0", + streamingTaskRunner ); final TaskLocation location1 = TaskLocation.create("testHost", 1234, -1); @@ -1595,6 +1624,9 @@ public Duration getEmissionDuration() supervisor.runInternal(); supervisor.handoffTaskGroupsEarly(ImmutableList.of(0)); + Assert.assertNull(id1.getCurrentRunnerStatus()); + Assert.assertEquals("NOT_STARTED", id1.getCurrentRunnerStatus()); + while (supervisor.getNoticesQueueSize() > 0) { Thread.sleep(100); } @@ -2704,6 +2736,8 @@ public String toString() private class TestSeekableStreamIndexTask extends SeekableStreamIndexTask { + private final SeekableStreamIndexTaskRunner streamingTaskRunner; + public TestSeekableStreamIndexTask( String id, @Nullable TaskResource taskResource, @@ -2713,6 +2747,29 @@ public TestSeekableStreamIndexTask( @Nullable Map context, @Nullable String groupId ) + { + this( + id, + taskResource, + dataSchema, + tuningConfig, + ioConfig, + context, + groupId, + null + ); + } + + public TestSeekableStreamIndexTask( + String id, + @Nullable TaskResource taskResource, + DataSchema dataSchema, + SeekableStreamIndexTaskTuningConfig tuningConfig, + SeekableStreamIndexTaskIOConfig ioConfig, + @Nullable Map context, + @Nullable String groupId, + @Nullable SeekableStreamIndexTaskRunner streamingTaskRunner + ) { super( id, @@ -2723,12 +2780,14 @@ public TestSeekableStreamIndexTask( context, groupId ); + this.streamingTaskRunner = streamingTaskRunner; } + @Nullable @Override protected SeekableStreamIndexTaskRunner createTaskRunner() { - return null; + return streamingTaskRunner; } @Override @@ -2799,7 +2858,7 @@ protected SeekableStreamIndexTaskIOConfig createTaskIoConfig( SeekableStreamSupervisorIOConfig ioConfig ) { - return new SeekableStreamIndexTaskIOConfig( + return new SeekableStreamIndexTaskIOConfig<>( groupId, baseSequenceName, new SeekableStreamStartSequenceNumbers<>(STREAM, startPartitions, exclusiveStartSequenceNumberPartitions), @@ -2871,7 +2930,7 @@ protected SeekableStreamDataSourceMetadata createDataSourceMetaD @Override protected OrderedSequenceNumber makeSequenceNumber(String seq, boolean isExclusive) { - return new OrderedSequenceNumber(seq, isExclusive) + return new OrderedSequenceNumber<>(seq, isExclusive) { @Override public int compareTo(OrderedSequenceNumber o) @@ -2905,7 +2964,7 @@ protected SeekableStreamSupervisorReportPayload createReportPayl boolean includeOffsets ) { - return new SeekableStreamSupervisorReportPayload( + return new SeekableStreamSupervisorReportPayload<>( DATASOURCE, STREAM, 1, @@ -3159,7 +3218,7 @@ private static SeekableStreamIndexTaskIOConfig createTaskIoConfigExt( SeekableStreamSupervisorIOConfig ioConfig ) { - return new SeekableStreamIndexTaskIOConfig( + return new SeekableStreamIndexTaskIOConfig<>( groupId, baseSequenceName, new SeekableStreamStartSequenceNumbers<>(STREAM, startPartitions, exclusiveStartSequenceNumberPartitions), diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/test/TestIndexerMetadataStorageCoordinator.java b/indexing-service/src/test/java/org/apache/druid/indexing/test/TestIndexerMetadataStorageCoordinator.java index 54e323581c47..a95d73ce1bb7 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/test/TestIndexerMetadataStorageCoordinator.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/test/TestIndexerMetadataStorageCoordinator.java @@ -36,7 +36,6 @@ import org.apache.druid.segment.realtime.appenderator.SegmentIdWithShardSpec; import org.apache.druid.timeline.DataSegment; import org.apache.druid.timeline.SegmentTimeline; -import org.apache.druid.timeline.partition.PartialShardSpec; import org.joda.time.DateTime; import org.joda.time.Interval; @@ -242,20 +241,16 @@ public int removeDataSourceMetadataOlderThan(long timestamp, @Nullable Set optional = injector.getInstance( - Key.get(new TypeLiteral>() {}) + Key.get(new TypeLiteral<>() {}) ); Assert.assertTrue(optional.isPresent()); } @@ -66,7 +66,7 @@ public void testGetShuffleMetricsWithNoShuffleMonitor() .thenReturn(Optional.empty()); final Injector injector = createInjector(monitorScheduler); final Optional optional = injector.getInstance( - Key.get(new TypeLiteral>() {}) + Key.get(new TypeLiteral<>() {}) ); Assert.assertFalse(optional.isPresent()); } diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/worker/shuffle/ShuffleResourceTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/worker/shuffle/ShuffleResourceTest.java index 69d48417deee..159800b4d4ee 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/worker/shuffle/ShuffleResourceTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/worker/shuffle/ShuffleResourceTest.java @@ -24,7 +24,6 @@ import com.google.common.util.concurrent.Futures; import com.google.common.util.concurrent.ListenableFuture; import org.apache.commons.io.FileUtils; -import org.apache.druid.client.indexing.NoopOverlordClient; import org.apache.druid.indexer.TaskState; import org.apache.druid.indexer.TaskStatus; import org.apache.druid.indexing.common.config.TaskConfig; @@ -32,6 +31,7 @@ import org.apache.druid.indexing.worker.config.WorkerConfig; import org.apache.druid.indexing.worker.shuffle.ShuffleMetrics.PerDatasourceShuffleMetrics; import org.apache.druid.java.util.common.Intervals; +import org.apache.druid.rpc.indexing.NoopOverlordClient; import org.apache.druid.rpc.indexing.OverlordClient; import org.apache.druid.segment.loading.StorageLocationConfig; import org.apache.druid.timeline.DataSegment; diff --git a/integration-tests-ex/cases/pom.xml b/integration-tests-ex/cases/pom.xml index 94c42a0555b4..f6c260603277 100644 --- a/integration-tests-ex/cases/pom.xml +++ b/integration-tests-ex/cases/pom.xml @@ -83,11 +83,6 @@ com.google.inject guice - - com.google.inject.extensions - guice-multibindings - provided - org.apache.curator curator-framework diff --git a/integration-tests-ex/cases/src/test/java/org/apache/druid/testsEx/catalog/ITCatalogIngestAndQueryTest.java b/integration-tests-ex/cases/src/test/java/org/apache/druid/testsEx/catalog/ITCatalogIngestAndQueryTest.java index 1a756910bf24..849ef1c4c2c0 100644 --- a/integration-tests-ex/cases/src/test/java/org/apache/druid/testsEx/catalog/ITCatalogIngestAndQueryTest.java +++ b/integration-tests-ex/cases/src/test/java/org/apache/druid/testsEx/catalog/ITCatalogIngestAndQueryTest.java @@ -30,8 +30,8 @@ import org.apache.druid.java.util.common.StringUtils; import org.apache.druid.java.util.common.logger.Logger; import org.apache.druid.query.QueryContexts; +import org.apache.druid.query.http.SqlTaskStatus; import org.apache.druid.sql.http.SqlQuery; -import org.apache.druid.sql.http.SqlTaskStatus; import org.apache.druid.testing.utils.DataLoaderHelper; import org.apache.druid.testing.utils.MsqTestQueryHelper; import org.apache.druid.testsEx.cluster.CatalogClient; diff --git a/integration-tests-ex/cases/src/test/java/org/apache/druid/testsEx/catalog/ITCatalogIngestErrorTest.java b/integration-tests-ex/cases/src/test/java/org/apache/druid/testsEx/catalog/ITCatalogIngestErrorTest.java index e075462d0bb6..fa5489dca80c 100644 --- a/integration-tests-ex/cases/src/test/java/org/apache/druid/testsEx/catalog/ITCatalogIngestErrorTest.java +++ b/integration-tests-ex/cases/src/test/java/org/apache/druid/testsEx/catalog/ITCatalogIngestErrorTest.java @@ -27,8 +27,8 @@ import org.apache.druid.catalog.model.table.DatasourceDefn; import org.apache.druid.catalog.model.table.TableBuilder; import org.apache.druid.java.util.common.StringUtils; +import org.apache.druid.query.http.SqlTaskStatus; import org.apache.druid.sql.http.SqlQuery; -import org.apache.druid.sql.http.SqlTaskStatus; import org.apache.druid.testing.utils.MsqTestQueryHelper; import org.apache.druid.testsEx.categories.Catalog; import org.apache.druid.testsEx.cluster.CatalogClient; diff --git a/integration-tests-ex/cases/src/test/java/org/apache/druid/testsEx/cluster/CatalogClient.java b/integration-tests-ex/cases/src/test/java/org/apache/druid/testsEx/cluster/CatalogClient.java index eee5b3d79730..03d79fef49d7 100644 --- a/integration-tests-ex/cases/src/test/java/org/apache/druid/testsEx/cluster/CatalogClient.java +++ b/integration-tests-ex/cases/src/test/java/org/apache/druid/testsEx/cluster/CatalogClient.java @@ -131,7 +131,7 @@ public List listSchemas() clusterClient.leadCoordinatorUrl(), CatalogResource.ROOT_PATH ); - return clusterClient.getAs(url, new TypeReference>() { }); + return clusterClient.getAs(url, new TypeReference<>() {}); } public List listTables() @@ -141,7 +141,7 @@ public List listTables() clusterClient.leadCoordinatorUrl(), CatalogResource.ROOT_PATH ); - return clusterClient.getAs(url, new TypeReference>() { }); + return clusterClient.getAs(url, new TypeReference<>() {}); } public List listTableNamesInSchema(String schemaName) @@ -152,7 +152,7 @@ public List listTableNamesInSchema(String schemaName) CatalogResource.ROOT_PATH, schemaName ); - return clusterClient.getAs(url, new TypeReference>() { }); + return clusterClient.getAs(url, new TypeReference<>() {}); } public List listTablesInSchema(String schemaName) @@ -163,6 +163,6 @@ public List listTablesInSchema(String schemaName) CatalogResource.ROOT_PATH, schemaName ); - return clusterClient.getAs(url, new TypeReference>() { }); + return clusterClient.getAs(url, new TypeReference<>() {}); } } diff --git a/integration-tests-ex/cases/src/test/java/org/apache/druid/testsEx/msq/ITKeyStatisticsSketchMergeMode.java b/integration-tests-ex/cases/src/test/java/org/apache/druid/testsEx/msq/ITKeyStatisticsSketchMergeMode.java index b0dee9e713eb..8fb325b1c651 100644 --- a/integration-tests-ex/cases/src/test/java/org/apache/druid/testsEx/msq/ITKeyStatisticsSketchMergeMode.java +++ b/integration-tests-ex/cases/src/test/java/org/apache/druid/testsEx/msq/ITKeyStatisticsSketchMergeMode.java @@ -24,8 +24,8 @@ import org.apache.druid.java.util.common.StringUtils; import org.apache.druid.msq.exec.ClusterStatisticsMergeMode; import org.apache.druid.msq.util.MultiStageQueryContext; +import org.apache.druid.query.http.SqlTaskStatus; import org.apache.druid.sql.http.SqlQuery; -import org.apache.druid.sql.http.SqlTaskStatus; import org.apache.druid.testing.clients.CoordinatorResourceTestClient; import org.apache.druid.testing.utils.DataLoaderHelper; import org.apache.druid.testing.utils.MsqTestQueryHelper; diff --git a/integration-tests-ex/cases/src/test/java/org/apache/druid/testsEx/msq/ITMultiStageQueryWorkerFaultTolerance.java b/integration-tests-ex/cases/src/test/java/org/apache/druid/testsEx/msq/ITMultiStageQueryWorkerFaultTolerance.java index 13c3fdc38463..5cc405045fd1 100644 --- a/integration-tests-ex/cases/src/test/java/org/apache/druid/testsEx/msq/ITMultiStageQueryWorkerFaultTolerance.java +++ b/integration-tests-ex/cases/src/test/java/org/apache/druid/testsEx/msq/ITMultiStageQueryWorkerFaultTolerance.java @@ -26,7 +26,7 @@ import org.apache.druid.java.util.common.StringUtils; import org.apache.druid.java.util.common.logger.Logger; import org.apache.druid.msq.util.MultiStageQueryContext; -import org.apache.druid.sql.http.SqlTaskStatus; +import org.apache.druid.query.http.SqlTaskStatus; import org.apache.druid.testing.clients.CoordinatorResourceTestClient; import org.apache.druid.testing.utils.DataLoaderHelper; import org.apache.druid.testing.utils.ITRetryUtil; diff --git a/integration-tests-ex/cases/src/test/java/org/apache/druid/testsEx/msq/MultiStageQuery.java b/integration-tests-ex/cases/src/test/java/org/apache/druid/testsEx/msq/MultiStageQuery.java index 3cdc57c37bfa..d77d5dfe2c99 100644 --- a/integration-tests-ex/cases/src/test/java/org/apache/druid/testsEx/msq/MultiStageQuery.java +++ b/integration-tests-ex/cases/src/test/java/org/apache/druid/testsEx/msq/MultiStageQuery.java @@ -29,7 +29,7 @@ import org.apache.druid.msq.indexing.report.MSQResultsReport; import org.apache.druid.msq.indexing.report.MSQTaskReport; import org.apache.druid.msq.indexing.report.MSQTaskReportPayload; -import org.apache.druid.sql.http.SqlTaskStatus; +import org.apache.druid.query.http.SqlTaskStatus; import org.apache.druid.storage.local.LocalFileExportStorageProvider; import org.apache.druid.testing.clients.CoordinatorResourceTestClient; import org.apache.druid.testing.utils.DataLoaderHelper; diff --git a/integration-tests-ex/cases/src/test/resources/indexer/export_task.json b/integration-tests-ex/cases/src/test/resources/indexer/export_task.json index 45e59704a85e..9cb87df9a4f3 100644 --- a/integration-tests-ex/cases/src/test/resources/indexer/export_task.json +++ b/integration-tests-ex/cases/src/test/resources/indexer/export_task.json @@ -198,7 +198,6 @@ }, "sqlResultsContext": { "timeZone": "UTC", - "serializeComplexValues": true, "stringifyArrays": false }, "sqlTypeNames": [ diff --git a/integration-tests-ex/tools/src/main/java/org/apache/druid/testing/tools/ServerManagerForQueryErrorTest.java b/integration-tests-ex/tools/src/main/java/org/apache/druid/testing/tools/ServerManagerForQueryErrorTest.java index d5b82e9c8d58..91434232b92c 100644 --- a/integration-tests-ex/tools/src/main/java/org/apache/druid/testing/tools/ServerManagerForQueryErrorTest.java +++ b/integration-tests-ex/tools/src/main/java/org/apache/druid/testing/tools/ServerManagerForQueryErrorTest.java @@ -147,7 +147,7 @@ protected QueryRunner buildQueryRunnerForSegment( return new ReportTimelineMissingSegmentQueryRunner<>(descriptor); } } else if (queryContext.getBoolean(QUERY_TIMEOUT_TEST_CONTEXT_KEY, false)) { - return (queryPlus, responseContext) -> new Sequence() + return (queryPlus, responseContext) -> new Sequence<>() { @Override public OutType accumulate(OutType initValue, Accumulator accumulator) @@ -162,7 +162,7 @@ public Yielder toYielder(OutType initValue, YieldingAccumulat } }; } else if (queryContext.getBoolean(QUERY_CAPACITY_EXCEEDED_TEST_CONTEXT_KEY, false)) { - return (queryPlus, responseContext) -> new Sequence() + return (queryPlus, responseContext) -> new Sequence<>() { @Override public OutType accumulate(OutType initValue, Accumulator accumulator) @@ -177,7 +177,7 @@ public Yielder toYielder(OutType initValue, YieldingAccumulat } }; } else if (queryContext.getBoolean(QUERY_UNSUPPORTED_TEST_CONTEXT_KEY, false)) { - return (queryPlus, responseContext) -> new Sequence() + return (queryPlus, responseContext) -> new Sequence<>() { @Override public OutType accumulate(OutType initValue, Accumulator accumulator) @@ -192,7 +192,7 @@ public Yielder toYielder(OutType initValue, YieldingAccumulat } }; } else if (queryContext.getBoolean(RESOURCE_LIMIT_EXCEEDED_TEST_CONTEXT_KEY, false)) { - return (queryPlus, responseContext) -> new Sequence() + return (queryPlus, responseContext) -> new Sequence<>() { @Override public OutType accumulate(OutType initValue, Accumulator accumulator) @@ -207,7 +207,7 @@ public Yielder toYielder(OutType initValue, YieldingAccumulat } }; } else if (queryContext.getBoolean(QUERY_FAILURE_TEST_CONTEXT_KEY, false)) { - return (queryPlus, responseContext) -> new Sequence() + return (queryPlus, responseContext) -> new Sequence<>() { @Override public OutType accumulate(OutType initValue, Accumulator accumulator) diff --git a/integration-tests/docker/Dockerfile b/integration-tests/docker/Dockerfile index df8c6f2598be..89fd45c97784 100644 --- a/integration-tests/docker/Dockerfile +++ b/integration-tests/docker/Dockerfile @@ -25,7 +25,17 @@ ARG KAFKA_VERSION # This is passed in by maven at build time to align with the client version we depend on in the pom file ARG ZK_VERSION ARG APACHE_ARCHIVE_MIRROR_HOST=https://archive.apache.org -RUN APACHE_ARCHIVE_MIRROR_HOST=${APACHE_ARCHIVE_MIRROR_HOST} /root/base-setup.sh && rm -f /root/base-setup.sh +ARG SETUP_RETRIES=3 +# Retry mechanism for running the setup script up to limit of 3 times. +RUN set -e; \ + for i in $(seq 1 $SETUP_RETRIES); do \ + echo "Attempt $i to run the setup script..."; \ + APACHE_ARCHIVE_MIRROR_HOST=${APACHE_ARCHIVE_MIRROR_HOST} /root/base-setup.sh && break || { \ + echo "Set up script attempt $i/$SETUP_RETRIES failed."; \ + sleep 2; \ + }; \ + done; \ + rm -f /root/base-setup.sh FROM druidbase ARG MYSQL_VERSION diff --git a/integration-tests/docker/docker-compose.high-availability.yml b/integration-tests/docker/docker-compose.high-availability.yml index 68049452d97c..0b38b90efaf0 100644 --- a/integration-tests/docker/docker-compose.high-availability.yml +++ b/integration-tests/docker/docker-compose.high-availability.yml @@ -132,7 +132,7 @@ services: - DRUID_INTEGRATION_TEST_GROUP=${DRUID_INTEGRATION_TEST_GROUP} - DRUID_SERVICE=custom-node-role - DRUID_LOG_PATH=/shared/logs/custom-node-role.log - - SERVICE_DRUID_JAVA_OPTS=-server -Xmx64m -Xms64m -XX:+UseG1GC -agentlib:jdwp=transport=dt_socket,server=y,suspend=n,address=5011 + - SERVICE_DRUID_JAVA_OPTS=-server -Xmx128m -Xms128m -XX:+UseG1GC -agentlib:jdwp=transport=dt_socket,server=y,suspend=n,address=5011 - druid_host=druid-custom-node-role - druid_auth_basic_common_cacheDirectory=/tmp/authCache/custom_node_role - druid_server_https_crlPath=/tls/revocations.crl diff --git a/integration-tests/docker/environment-configs/router-custom-check-tls b/integration-tests/docker/environment-configs/router-custom-check-tls index 0414b16b65c9..83ba28fad7a2 100644 --- a/integration-tests/docker/environment-configs/router-custom-check-tls +++ b/integration-tests/docker/environment-configs/router-custom-check-tls @@ -21,7 +21,7 @@ DRUID_SERVICE=router DRUID_LOG_PATH=/shared/logs/router-custom-check-tls.log # JAVA OPTS -SERVICE_DRUID_JAVA_OPTS=-server -Xmx64m -Xms64m -XX:+UseG1GC -XX:+PrintGCDetails -agentlib:jdwp=transport=dt_socket,server=y,suspend=n,address=5003 +SERVICE_DRUID_JAVA_OPTS=-server -Xmx128m -Xms128m -XX:+UseG1GC -XX:+PrintGCDetails -agentlib:jdwp=transport=dt_socket,server=y,suspend=n,address=5003 # Druid configs druid_plaintextPort=8891 diff --git a/integration-tests/docker/environment-configs/router-no-client-auth-tls b/integration-tests/docker/environment-configs/router-no-client-auth-tls index 9396b3b17424..f22226cd423e 100644 --- a/integration-tests/docker/environment-configs/router-no-client-auth-tls +++ b/integration-tests/docker/environment-configs/router-no-client-auth-tls @@ -21,7 +21,7 @@ DRUID_SERVICE=router DRUID_LOG_PATH=/shared/logs/router-no-client-auth-tls.log # JAVA OPTS -SERVICE_DRUID_JAVA_OPTS=-server -Xmx64m -Xms64m -XX:+UseG1GC -XX:+PrintGCDetails -agentlib:jdwp=transport=dt_socket,server=y,suspend=n,address=5002 +SERVICE_DRUID_JAVA_OPTS=-server -Xmx128m -Xms128m -XX:+UseG1GC -XX:+PrintGCDetails -agentlib:jdwp=transport=dt_socket,server=y,suspend=n,address=5002 # Druid configs druid_plaintextPort=8890 diff --git a/integration-tests/docker/environment-configs/router-permissive-tls b/integration-tests/docker/environment-configs/router-permissive-tls index d827dae555bc..9cd87fb3660e 100644 --- a/integration-tests/docker/environment-configs/router-permissive-tls +++ b/integration-tests/docker/environment-configs/router-permissive-tls @@ -21,7 +21,7 @@ DRUID_SERVICE=router DRUID_LOG_PATH=/shared/logs/router-permissive-tls.log # JAVA OPTS -SERVICE_DRUID_JAVA_OPTS=-server -Xmx64m -Xms64m -XX:+UseG1GC -XX:+PrintGCDetails -agentlib:jdwp=transport=dt_socket,server=y,suspend=n,address=5001 +SERVICE_DRUID_JAVA_OPTS=-server -Xmx128m -Xms128m -XX:+UseG1GC -XX:+PrintGCDetails -agentlib:jdwp=transport=dt_socket,server=y,suspend=n,address=5001 # Druid configs druid_plaintextPort=8889 diff --git a/integration-tests/pom.xml b/integration-tests/pom.xml index 7bea258c6472..411ba0a2d250 100644 --- a/integration-tests/pom.xml +++ b/integration-tests/pom.xml @@ -278,11 +278,7 @@ com.google.inject guice - - com.google.inject.extensions - guice-multibindings - provided - + com.fasterxml.jackson.core jackson-databind @@ -666,7 +662,7 @@ ${jdk.strong.encapsulation.argLine} - -Xmx128m + -Xmx256m -Duser.timezone=UTC -Dfile.encoding=UTF-8 -Ddruid.test.config.dockerIp=${env.DOCKER_IP} diff --git a/integration-tests/src/main/java/org/apache/druid/server/coordination/ServerManagerForQueryErrorTest.java b/integration-tests/src/main/java/org/apache/druid/server/coordination/ServerManagerForQueryErrorTest.java index 8bfceb544708..6e225304a4a3 100644 --- a/integration-tests/src/main/java/org/apache/druid/server/coordination/ServerManagerForQueryErrorTest.java +++ b/integration-tests/src/main/java/org/apache/druid/server/coordination/ServerManagerForQueryErrorTest.java @@ -145,7 +145,7 @@ protected QueryRunner buildQueryRunnerForSegment( return new ReportTimelineMissingSegmentQueryRunner<>(descriptor); } } else if (queryContext.getBoolean(QUERY_TIMEOUT_TEST_CONTEXT_KEY, false)) { - return (queryPlus, responseContext) -> new Sequence() + return (queryPlus, responseContext) -> new Sequence<>() { @Override public OutType accumulate(OutType initValue, Accumulator accumulator) @@ -160,7 +160,7 @@ public Yielder toYielder(OutType initValue, YieldingAccumulat } }; } else if (queryContext.getBoolean(QUERY_CAPACITY_EXCEEDED_TEST_CONTEXT_KEY, false)) { - return (queryPlus, responseContext) -> new Sequence() + return (queryPlus, responseContext) -> new Sequence<>() { @Override public OutType accumulate(OutType initValue, Accumulator accumulator) @@ -175,7 +175,7 @@ public Yielder toYielder(OutType initValue, YieldingAccumulat } }; } else if (queryContext.getBoolean(QUERY_UNSUPPORTED_TEST_CONTEXT_KEY, false)) { - return (queryPlus, responseContext) -> new Sequence() + return (queryPlus, responseContext) -> new Sequence<>() { @Override public OutType accumulate(OutType initValue, Accumulator accumulator) @@ -190,7 +190,7 @@ public Yielder toYielder(OutType initValue, YieldingAccumulat } }; } else if (queryContext.getBoolean(RESOURCE_LIMIT_EXCEEDED_TEST_CONTEXT_KEY, false)) { - return (queryPlus, responseContext) -> new Sequence() + return (queryPlus, responseContext) -> new Sequence<>() { @Override public OutType accumulate(OutType initValue, Accumulator accumulator) @@ -205,7 +205,7 @@ public Yielder toYielder(OutType initValue, YieldingAccumulat } }; } else if (queryContext.getBoolean(QUERY_FAILURE_TEST_CONTEXT_KEY, false)) { - return (queryPlus, responseContext) -> new Sequence() + return (queryPlus, responseContext) -> new Sequence<>() { @Override public OutType accumulate(OutType initValue, Accumulator accumulator) diff --git a/integration-tests/src/main/java/org/apache/druid/testing/clients/AbstractQueryResourceTestClient.java b/integration-tests/src/main/java/org/apache/druid/testing/clients/AbstractQueryResourceTestClient.java index c558ff91ca3d..18a8affeac7a 100644 --- a/integration-tests/src/main/java/org/apache/druid/testing/clients/AbstractQueryResourceTestClient.java +++ b/integration-tests/src/main/java/org/apache/druid/testing/clients/AbstractQueryResourceTestClient.java @@ -99,9 +99,7 @@ public byte[] encode(Object content) throws IOException @Override public List> decode(byte[] content) throws IOException { - return om.readValue(content, new TypeReference>>() - { - }); + return om.readValue(content, new TypeReference<>() {}); } } diff --git a/integration-tests/src/main/java/org/apache/druid/testing/clients/ClientInfoResourceTestClient.java b/integration-tests/src/main/java/org/apache/druid/testing/clients/ClientInfoResourceTestClient.java index 09dfff04c605..0a54ee6e581f 100644 --- a/integration-tests/src/main/java/org/apache/druid/testing/clients/ClientInfoResourceTestClient.java +++ b/integration-tests/src/main/java/org/apache/druid/testing/clients/ClientInfoResourceTestClient.java @@ -89,9 +89,7 @@ public List getDimensions(String dataSource, String interval) ); } return jsonMapper.readValue( - response.getContent(), new TypeReference>() - { - } + response.getContent(), new TypeReference<>() {} ); } catch (Exception e) { diff --git a/integration-tests/src/main/java/org/apache/druid/testing/clients/CompactionResourceTestClient.java b/integration-tests/src/main/java/org/apache/druid/testing/clients/CompactionResourceTestClient.java index 74d56065ef45..849af4975673 100644 --- a/integration-tests/src/main/java/org/apache/druid/testing/clients/CompactionResourceTestClient.java +++ b/integration-tests/src/main/java/org/apache/druid/testing/clients/CompactionResourceTestClient.java @@ -113,7 +113,7 @@ public DruidCompactionConfig getCompactionConfig() throws Exception response.getContent() ); } - return jsonMapper.readValue(response.getContent(), new TypeReference() {}); + return jsonMapper.readValue(response.getContent(), new TypeReference<>() {}); } public DataSourceCompactionConfig getDataSourceCompactionConfig(String dataSource) throws Exception @@ -129,7 +129,7 @@ public DataSourceCompactionConfig getDataSourceCompactionConfig(String dataSourc response.getContent() ); } - return jsonMapper.readValue(response.getContent(), new TypeReference() {}); + return jsonMapper.readValue(response.getContent(), new TypeReference<>() {}); } public void forceTriggerAutoCompaction() throws Exception @@ -183,7 +183,7 @@ public Map getCompactionProgress(String dataSource) throws Excep response.getContent() ); } - return jsonMapper.readValue(response.getContent(), new TypeReference>() {}); + return jsonMapper.readValue(response.getContent(), new TypeReference<>() {}); } public Map getCompactionStatus(String dataSource) throws Exception @@ -201,7 +201,7 @@ public Map getCompactionStatus(String dataSource) throws Excepti response.getContent() ); } - Map>> latestSnapshots = jsonMapper.readValue(response.getContent(), new TypeReference>>>() {}); + Map>> latestSnapshots = jsonMapper.readValue(response.getContent(), new TypeReference<>() {}); return latestSnapshots.get("latestStatus").get(0); } } diff --git a/integration-tests/src/main/java/org/apache/druid/testing/clients/CoordinatorResourceTestClient.java b/integration-tests/src/main/java/org/apache/druid/testing/clients/CoordinatorResourceTestClient.java index 7a42119e6cca..e82580cd312e 100644 --- a/integration-tests/src/main/java/org/apache/druid/testing/clients/CoordinatorResourceTestClient.java +++ b/integration-tests/src/main/java/org/apache/druid/testing/clients/CoordinatorResourceTestClient.java @@ -131,9 +131,7 @@ public List getSegments(final String dataSource) ); segments = jsonMapper.readValue( - response.getContent(), new TypeReference>() - { - } + response.getContent(), new TypeReference<>() {} ); } catch (Exception e) { @@ -152,9 +150,7 @@ public List getFullSegmentsMetadata(final String dataSource) ); segments = jsonMapper.readValue( - response.getContent(), new TypeReference>() - { - } + response.getContent(), new TypeReference<>() {} ); } catch (Exception e) { @@ -171,9 +167,7 @@ public List getSegmentIntervals(final String dataSource) StatusResponseHolder response = makeRequest(HttpMethod.GET, getIntervalsURL(dataSource)); segments = jsonMapper.readValue( - response.getContent(), new TypeReference>() - { - } + response.getContent(), new TypeReference<>() {} ); } catch (Exception e) { @@ -189,9 +183,7 @@ public List getAvailableSegments(final String dataSource) StatusResponseHolder response = makeRequest(HttpMethod.GET, getFullSegmentsURL(dataSource)); return jsonMapper.readValue( - response.getContent(), new TypeReference>() - { - } + response.getContent(), new TypeReference<>() {} ); } catch (Exception e) { @@ -222,9 +214,7 @@ private Map getLoadStatus(String dataSource) } status = jsonMapper.readValue( - response.getContent(), new TypeReference>() - { - } + response.getContent(), new TypeReference<>() {} ); } catch (Exception e) { @@ -333,9 +323,7 @@ public Map initializeLookups(String filePath) throws Exception Map results2 = jsonMapper.readValue( response.getContent(), - new TypeReference>() - { - } + new TypeReference<>() {} ); return results2; @@ -367,9 +355,7 @@ private Map>>>() - { - } + new TypeReference<>() {} ); } catch (Exception e) { diff --git a/integration-tests/src/main/java/org/apache/druid/testing/clients/OverlordResourceTestClient.java b/integration-tests/src/main/java/org/apache/druid/testing/clients/OverlordResourceTestClient.java index f75dc6043f9d..9b6c3ec1ef74 100644 --- a/integration-tests/src/main/java/org/apache/druid/testing/clients/OverlordResourceTestClient.java +++ b/integration-tests/src/main/java/org/apache/druid/testing/clients/OverlordResourceTestClient.java @@ -147,9 +147,7 @@ public TaskStatusPlus getTaskStatus(String taskID) LOG.debug("Index status response" + response.getContent()); TaskStatusResponse taskStatusResponse = jsonMapper.readValue( response.getContent(), - new TypeReference() - { - } + new TypeReference<>() {} ); return taskStatusResponse.getStatus(); } @@ -236,9 +234,7 @@ private List getTasks(String identifier) LOG.debug("Tasks %s response %s", identifier, response.getContent()); } return jsonMapper.readValue( - response.getContent(), new TypeReference>() - { - } + response.getContent(), new TypeReference<>() {} ); } catch (Exception e) { @@ -257,9 +253,7 @@ public TaskPayloadResponse getTaskPayload(String taskId) LOG.debug("Task %s response %s", taskId, response.getContent()); } return jsonMapper.readValue( - response.getContent(), new TypeReference() - { - } + response.getContent(), new TypeReference<>() {} ); } catch (ISE e) { @@ -350,9 +344,7 @@ public Map> getLockedIntervals(List loc ).get(); return jsonMapper.readValue( response.getContent(), - new TypeReference>>() - { - } + new TypeReference<>() {} ); } catch (Exception e) { @@ -368,7 +360,7 @@ public void waitUntilTaskCompletes(final String taskID) public void waitUntilTaskCompletes(final String taskID, final long millisEach, final int numTimes) { ITRetryUtil.retryUntil( - new Callable() + new Callable<>() { @Override public Boolean call() @@ -398,7 +390,7 @@ public void waitUntilTaskFails(final String taskID) public void waitUntilTaskFails(final String taskID, final long millisEach, final int numTimes) { ITRetryUtil.retryUntil( - new Callable() + new Callable<>() { @Override public Boolean call() @@ -758,9 +750,7 @@ public List getSupervisorHistory(String id) ); } List responseData = jsonMapper.readValue( - response.getContent(), new TypeReference>() - { - } + response.getContent(), new TypeReference<>() {} ); return responseData; } diff --git a/integration-tests/src/main/java/org/apache/druid/testing/utils/AbstractTestQueryHelper.java b/integration-tests/src/main/java/org/apache/druid/testing/utils/AbstractTestQueryHelper.java index f680ad909a63..673db41e7f74 100644 --- a/integration-tests/src/main/java/org/apache/druid/testing/utils/AbstractTestQueryHelper.java +++ b/integration-tests/src/main/java/org/apache/druid/testing/utils/AbstractTestQueryHelper.java @@ -117,9 +117,7 @@ public void testQueriesFromFile(String url, String filePath) throws Exception List queries = jsonMapper.readValue( TestQueryHelper.class.getResourceAsStream(filePath), - new TypeReference>() - { - } + new TypeReference<>() {} ); testQueries(url, queries); @@ -131,9 +129,7 @@ public void testQueriesFromString(String url, String str) throws Exception List queries = jsonMapper.readValue( str, - new TypeReference>() - { - } + new TypeReference<>() {} ); testQueries(url, queries); } diff --git a/integration-tests/src/main/java/org/apache/druid/testing/utils/MsqTestQueryHelper.java b/integration-tests/src/main/java/org/apache/druid/testing/utils/MsqTestQueryHelper.java index 2a4d04f42dbd..0b544244509a 100644 --- a/integration-tests/src/main/java/org/apache/druid/testing/utils/MsqTestQueryHelper.java +++ b/integration-tests/src/main/java/org/apache/druid/testing/utils/MsqTestQueryHelper.java @@ -37,8 +37,8 @@ import org.apache.druid.msq.indexing.report.MSQResultsReport; import org.apache.druid.msq.indexing.report.MSQTaskReport; import org.apache.druid.msq.indexing.report.MSQTaskReportPayload; +import org.apache.druid.query.http.SqlTaskStatus; import org.apache.druid.sql.http.SqlQuery; -import org.apache.druid.sql.http.SqlTaskStatus; import org.apache.druid.testing.IntegrationTestingConfig; import org.apache.druid.testing.clients.OverlordResourceTestClient; import org.apache.druid.testing.clients.SqlResourceTestClient; @@ -269,9 +269,7 @@ public void testQueriesFromFile(String filePath, String fullDatasourcePath) thro List queries = jsonMapper.readValue( TestQueryHelper.class.getResourceAsStream(filePath), - new TypeReference>() - { - } + new TypeReference<>() {} ); for (MsqQueryWithResults queryWithResults : queries) { String queryString = queryWithResults.getQuery(); diff --git a/integration-tests/src/main/java/org/apache/druid/testing/utils/ServerDiscoveryUtil.java b/integration-tests/src/main/java/org/apache/druid/testing/utils/ServerDiscoveryUtil.java index 56b475f102fa..cfaa7080b21b 100644 --- a/integration-tests/src/main/java/org/apache/druid/testing/utils/ServerDiscoveryUtil.java +++ b/integration-tests/src/main/java/org/apache/druid/testing/utils/ServerDiscoveryUtil.java @@ -50,7 +50,7 @@ public static boolean isInstanceReady(ServerDiscoverySelector serviceProvider) public static void waitUntilInstanceReady(final ServerDiscoverySelector serviceProvider, String instanceType) { ITRetryUtil.retryUntilTrue( - new Callable() + new Callable<>() { @Override public Boolean call() diff --git a/integration-tests/src/test/java/org/apache/druid/tests/coordinator/duty/ITAutoCompactionTest.java b/integration-tests/src/test/java/org/apache/druid/tests/coordinator/duty/ITAutoCompactionTest.java index 7895423268a9..8218328c9221 100644 --- a/integration-tests/src/test/java/org/apache/druid/tests/coordinator/duty/ITAutoCompactionTest.java +++ b/integration-tests/src/test/java/org/apache/druid/tests/coordinator/duty/ITAutoCompactionTest.java @@ -149,7 +149,7 @@ public void testAutoCompactionRowWithMetricAndRowWithoutMetricShouldPreserveExis intervalsBeforeCompaction.sort(null); // 2 segments across 1 days... verifySegmentsCount(2); - ArrayList nullList = new ArrayList(); + ArrayList nullList = new ArrayList<>(); nullList.add(null); Map queryAndResultFields = ImmutableMap.of( "%%FIELD_TO_QUERY%%", "added", @@ -245,7 +245,7 @@ public void testAutoCompactionRowWithMetricAndRowWithoutMetricShouldPreserveExis intervalsBeforeCompaction.sort(null); // 2 segments across 1 days... verifySegmentsCount(2); - ArrayList nullList = new ArrayList(); + ArrayList nullList = new ArrayList<>(); nullList.add(null); Map queryAndResultFields = ImmutableMap.of( "%%FIELD_TO_QUERY%%", "added", @@ -348,7 +348,7 @@ public void testAutoCompactionOnlyRowsWithoutMetricShouldAddNewMetrics() throws intervalsBeforeCompaction.sort(null); // 2 segments across 1 days... verifySegmentsCount(2); - ArrayList nullList = new ArrayList(); + ArrayList nullList = new ArrayList<>(); nullList.add(null); Map queryAndResultFields = ImmutableMap.of( "%%FIELD_TO_QUERY%%", "added", diff --git a/integration-tests/src/test/java/org/apache/druid/tests/query/ITQueryRetryTestOnMissingSegments.java b/integration-tests/src/test/java/org/apache/druid/tests/query/ITQueryRetryTestOnMissingSegments.java index 999ebc8c6804..f05e5c42e6e8 100644 --- a/integration-tests/src/test/java/org/apache/druid/tests/query/ITQueryRetryTestOnMissingSegments.java +++ b/integration-tests/src/test/java/org/apache/druid/tests/query/ITQueryRetryTestOnMissingSegments.java @@ -132,9 +132,7 @@ private void testQueries(String queryWithResultsStr, Expectation expectation) th { final List queries = jsonMapper.readValue( queryWithResultsStr, - new TypeReference>() - { - } + new TypeReference<>() {} ); testQueries(queries, expectation); } @@ -156,9 +154,7 @@ private void testQueries(List queries, Expectation expectation List> result = jsonMapper.readValue( responseHolder.getContent(), - new TypeReference>>() - { - } + new TypeReference<>() {} ); if (!QueryResultVerifier.compareResults( result, diff --git a/integration-tests/src/test/java/org/apache/druid/tests/security/AbstractAuthConfigurationTest.java b/integration-tests/src/test/java/org/apache/druid/tests/security/AbstractAuthConfigurationTest.java index 1c5867916137..b2ecc596e7c6 100644 --- a/integration-tests/src/test/java/org/apache/druid/tests/security/AbstractAuthConfigurationTest.java +++ b/integration-tests/src/test/java/org/apache/druid/tests/security/AbstractAuthConfigurationTest.java @@ -94,9 +94,7 @@ public abstract class AbstractAuthConfigurationTest "SELECT * FROM sys.tasks WHERE datasource IN ('auth_test')"; protected static final TypeReference>> SYS_SCHEMA_RESULTS_TYPE_REFERENCE = - new TypeReference>>() - { - }; + new TypeReference<>() {}; /** * create a ResourceAction set of permissions that can only read a 'auth_test' datasource, for Authorizer diff --git a/integration-tests/src/test/resources/queries/twitterstream_queries.json b/integration-tests/src/test/resources/queries/twitterstream_queries.json index cdd4057eb572..104b07ba47a2 100644 --- a/integration-tests/src/test/resources/queries/twitterstream_queries.json +++ b/integration-tests/src/test/resources/queries/twitterstream_queries.json @@ -94,7 +94,8 @@ "context": { "useCache": "true", "populateCache": "true", - "timeout": 60000 + "timeout": 60000, + "useTopNMultiPassPooledQueryGranularity": "true" } }, "expectedResults": [ @@ -198,7 +199,8 @@ "context": { "useCache": "true", "populateCache": "true", - "timeout": 60000 + "timeout": 60000, + "useTopNMultiPassPooledQueryGranularity": "true" } }, "expectedResults": [ @@ -322,7 +324,8 @@ "context": { "useCache": "true", "populateCache": "true", - "timeout": 60000 + "timeout": 60000, + "useTopNMultiPassPooledQueryGranularity": "true" } }, "expectedResults": [ @@ -741,7 +744,8 @@ "context": { "useCache": "true", "populateCache": "true", - "timeout": 60000 + "timeout": 60000, + "useTopNMultiPassPooledQueryGranularity": "true" } }, "expectedResults": [ diff --git a/it.sh b/it.sh index 12519c24b54d..cf9e02e2b2cc 100755 --- a/it.sh +++ b/it.sh @@ -229,7 +229,7 @@ case $CMD in usage ;; "ci" ) - mvn -q clean install dependency:go-offline -P dist $MAVEN_IGNORE + mvn -q clean install dependency:go-offline -P dist $MAVEN_IGNORE -T1C ;; "build" ) mvn -B clean install -P dist $MAVEN_IGNORE -T1.0C $* diff --git a/licenses.yaml b/licenses.yaml index 9b11d2a6e294..e3f822300cfb 100644 --- a/licenses.yaml +++ b/licenses.yaml @@ -371,19 +371,15 @@ name: Guice license_category: binary module: java-core license_name: Apache License version 2.0 -version: 4.2.2 +version: 5.1.0 libraries: - com.google.inject: guice - - com.google.inject.extensions: guice-multibindings - com.google.inject.extensions: guice-servlet - com.google.inject.extensions: guice-assistedinject notices: - guice: | Google Guice - Core Library Copyright 2006-2016 Google, Inc. - - guice-multibindings: | - Google Guice - Extensions - MultiBindings - Copyright 2006-2016 Google, Inc. - guice-servlet: | Google Guice - Extensions - Servlet Copyright 2006-2016 Google, Inc. @@ -3165,7 +3161,7 @@ libraries: --- name: Apache Kafka -version: 3.6.1 +version: 3.9.0 license_category: binary module: extensions/druid-kafka-indexing-service license_name: Apache License version 2.0 @@ -3174,7 +3170,7 @@ libraries: notices: - kafka-clients: | Apache Kafka - Copyright 2023 The Apache Software Foundation. + Copyright 2024 The Apache Software Foundation. This product includes software developed at The Apache Software Foundation (https://www.apache.org/). @@ -5125,15 +5121,6 @@ version: 5.2.5 --- -name: "@druid-toolkit/query" -license_category: binary -module: web-console -license_name: Apache License version 2.0 -copyright: Imply Data -version: 0.22.23 - ---- - name: "@emotion/cache" license_category: binary module: web-console @@ -5224,6 +5211,16 @@ license_file_path: licenses/bin/@emotion-weak-memoize.MIT --- +name: "@flatten-js/interval-tree" +license_category: binary +module: web-console +license_name: MIT License +copyright: Alex Bol +version: 1.1.3 +license_file_path: licenses/bin/@flatten-js-interval-tree.MIT + +--- + name: "@fontsource/open-sans" license_category: binary module: web-console @@ -5234,6 +5231,15 @@ license_file_path: licenses/bin/@fontsource-open-sans.OFL --- +name: "@internationalized/date" +license_category: binary +module: web-console +license_name: Apache License version 2.0 +copyright: Adobe +version: 3.5.6 + +--- + name: "@popperjs/core" license_category: binary module: web-console @@ -5244,6 +5250,15 @@ license_file_path: licenses/bin/@popperjs-core.MIT --- +name: "@swc/helpers" +license_category: binary +module: web-console +license_name: Apache License version 2.0 +copyright: 강동윤 +version: 0.5.13 + +--- + name: "@types/parse-json" license_category: binary module: web-console @@ -5404,15 +5419,6 @@ license_file_path: licenses/bin/change-case.MIT --- -name: "chronoshift" -license_category: binary -module: web-console -license_name: Apache License version 2.0 -copyright: Vadim Ogievetsky -version: 0.10.0 - ---- - name: "classnames" license_category: binary module: web-console @@ -5702,6 +5708,15 @@ license_file_path: licenses/bin/dot-case.MIT --- +name: "druid-query-toolkit" +license_category: binary +module: web-console +license_name: Apache License version 2.0 +copyright: Imply Data +version: 1.0.0 + +--- + name: "echarts" license_category: binary module: web-console @@ -5801,16 +5816,6 @@ license_file_path: licenses/bin/has-flag.MIT --- -name: "has-own-prop" -license_category: binary -module: web-console -license_name: MIT License -copyright: Sindre Sorhus -version: 2.0.0 -license_file_path: licenses/bin/has-own-prop.MIT - ---- - name: "hasown" license_category: binary module: web-console @@ -5871,15 +5876,6 @@ license_file_path: licenses/bin/iconv-lite.MIT --- -name: "immutable-class" -license_category: binary -module: web-console -license_name: Apache License version 2.0 -copyright: Vadim Ogievetsky -version: 0.11.2 - ---- - name: "import-fresh" license_category: binary module: web-console @@ -6060,26 +6056,6 @@ license_file_path: licenses/bin/mime-types.MIT --- -name: "moment-timezone" -license_category: binary -module: web-console -license_name: MIT License -copyright: Tim Wood -version: 0.5.43 -license_file_path: licenses/bin/moment-timezone.MIT - ---- - -name: "moment" -license_category: binary -module: web-console -license_name: MIT License -copyright: Iskren Ivov Chernev -version: 2.29.4 -license_file_path: licenses/bin/moment.MIT - ---- - name: "no-case" license_category: binary module: web-console diff --git a/owasp-dependency-check-suppressions.xml b/owasp-dependency-check-suppressions.xml index 8168a2cf58c6..3c634a0df358 100644 --- a/owasp-dependency-check-suppressions.xml +++ b/owasp-dependency-check-suppressions.xml @@ -182,13 +182,15 @@ + ^pkg:maven/org\.apache\.kafka/kafka\-clients@.*$ CVE-2022-34917 CVE-2023-25194 + CVE-2024-31141 diff --git a/pom.xml b/pom.xml index b5f48ffbb2a2..c67df8127a03 100644 --- a/pom.xml +++ b/pom.xml @@ -75,7 +75,7 @@ UTF-8 0.9.0.M2 5.5.0 - 3.6.1 + 3.9.0 2.4.0 @@ -95,7 +95,7 @@ 2.35.1 8.5.4 32.0.1-jre - 4.2.2 + 5.1.0 1.3 9.4.56.v20240826 1.19.4 @@ -596,11 +596,6 @@ guice-servlet ${guice.version} - - com.google.inject.extensions - guice-multibindings - ${guice.version} - com.google.inject.extensions guice-assistedinject @@ -1403,6 +1398,18 @@ joni 2.1.34 + + org.testcontainers + testcontainers + 1.16.3 + test + + + org.testcontainers + junit-jupiter + 1.16.3 + test + @@ -1654,6 +1661,9 @@ sun.misc.Unsafe java.lang.invoke.MethodHandle + + java.lang.ClassLoader