From ca0e6996d265cb73466a9c07e304416c62541234 Mon Sep 17 00:00:00 2001 From: Kashif Faraz Date: Mon, 4 Aug 2025 13:07:47 +0530 Subject: [PATCH 01/10] Migrate revised security IT to embedded tests --- .github/workflows/revised-its.yml | 2 +- embedded-tests/pom.xml | 6 + .../embedded/auth/BasicAuthIndexingTest.java | 46 +++ .../embedded/auth/BasicAuthMsqTest.java | 353 +++++++++++++----- .../testing/embedded/auth/SecurityClient.java | 177 +++++++++ .../embedded/compact/AutoCompactionTest.java | 10 +- .../compact/AutoCompactionUpgradeTest.java | 10 +- .../compact/CompactionResourceTestClient.java | 264 ++++--------- .../compact/CompactionSparseColumnTest.java | 4 +- .../embedded/compact/CompactionTaskTest.java | 6 +- .../indexing/IndexParallelTaskTest.java | 8 +- .../embedded/indexing/MoreResources.java | 6 +- .../embedded/msq/BaseRealtimeQueryTest.java | 5 +- .../testing/embedded/msq/EmbeddedMSQApis.java | 20 +- .../embedded/msq/MsqExportDirectory.java | 48 ++- .../embedded/server/OverlordClientTest.java | 52 +-- .../indexing/common/task/TaskBuilder.java | 26 +- .../cases/cluster/Security/docker-compose.py | 36 -- integration-tests-ex/cases/pom.xml | 9 - .../druid/testsEx/categories/Security.java | 24 -- .../test/resources/indexer/export_task.json | 222 ----------- .../druid/testing/clients/SecurityClient.java | 238 ------------ .../coordinator/CoordinatorServiceClient.java | 14 +- .../druid/rpc/guice/ServiceClientModule.java | 84 +++-- .../testing/embedded/EmbeddedClusterApis.java | 52 ++- .../embedded/EmbeddedDruidCluster.java | 40 +- .../embedded/EmbeddedServerLifecycle.java | 2 +- .../embedded/EmbeddedServiceClient.java | 218 +++++++++++ .../embedded/ServerReferenceHolder.java | 23 +- .../embedded/ServerReferencesProvider.java | 9 + .../testing/embedded/indexing/Resources.java | 42 ++- 31 files changed, 1028 insertions(+), 1028 deletions(-) create mode 100644 embedded-tests/src/test/java/org/apache/druid/testing/embedded/auth/BasicAuthIndexingTest.java rename integration-tests-ex/cases/src/test/java/org/apache/druid/testsEx/auth/ITSecurityBasicQuery.java => embedded-tests/src/test/java/org/apache/druid/testing/embedded/auth/BasicAuthMsqTest.java (53%) create mode 100644 embedded-tests/src/test/java/org/apache/druid/testing/embedded/auth/SecurityClient.java rename services/src/test/java/org/apache/druid/testing/embedded/ClusterReferencesProvider.java => embedded-tests/src/test/java/org/apache/druid/testing/embedded/msq/MsqExportDirectory.java (52%) delete mode 100644 integration-tests-ex/cases/cluster/Security/docker-compose.py delete mode 100644 integration-tests-ex/cases/src/test/java/org/apache/druid/testsEx/categories/Security.java delete mode 100644 integration-tests-ex/cases/src/test/resources/indexer/export_task.json delete mode 100644 integration-tests/src/main/java/org/apache/druid/testing/clients/SecurityClient.java create mode 100644 services/src/test/java/org/apache/druid/testing/embedded/EmbeddedServiceClient.java diff --git a/.github/workflows/revised-its.yml b/.github/workflows/revised-its.yml index 148f04f4b7f8..575f28c25acf 100644 --- a/.github/workflows/revised-its.yml +++ b/.github/workflows/revised-its.yml @@ -67,7 +67,7 @@ jobs: fail-fast: false matrix: jdk: [17] - it: [HighAvailability, MultiStageQuery, Catalog, BatchIndex, MultiStageQueryWithMM, InputSource, InputFormat, Security, Query, DruidExactCountBitmap] + it: [HighAvailability, MultiStageQuery, Catalog, BatchIndex, MultiStageQueryWithMM, InputSource, InputFormat, Query, DruidExactCountBitmap] indexer: [middleManager] uses: ./.github/workflows/reusable-revised-its.yml if: ${{ needs.changes.outputs.core == 'true' || needs.changes.outputs.common-extensions == 'true' }} diff --git a/embedded-tests/pom.xml b/embedded-tests/pom.xml index a69976e1f9e9..bf8840ecd47b 100644 --- a/embedded-tests/pom.xml +++ b/embedded-tests/pom.xml @@ -86,6 +86,12 @@ ${project.parent.version} test + + org.apache.druid.extensions + druid-basic-security + ${project.parent.version} + test + joda-time joda-time diff --git a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/auth/BasicAuthIndexingTest.java b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/auth/BasicAuthIndexingTest.java new file mode 100644 index 000000000000..19f5d14ad6b8 --- /dev/null +++ b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/auth/BasicAuthIndexingTest.java @@ -0,0 +1,46 @@ +/* + * 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.testing.embedded.auth; + +import org.apache.druid.security.basic.BasicSecurityDruidModule; +import org.apache.druid.testing.embedded.EmbeddedDruidCluster; +import org.apache.druid.testing.embedded.indexing.IndexTaskTest; + +public class BasicAuthIndexingTest extends IndexTaskTest +{ + @Override + public EmbeddedDruidCluster createCluster() + { + return super + .createCluster() + .addExtension(BasicSecurityDruidModule.class) + .addCommonProperty("druid.auth.authenticatorChain", "[\"basic\"]") + .addCommonProperty("druid.auth.authenticator.basic.type", "basic") + .addCommonProperty("druid.auth.authenticator.basic.initialAdminPassword", "priest") + .addCommonProperty("druid.auth.authenticator.basic.initialInternalClientPassword", "warlock") + .addCommonProperty("druid.auth.authenticator.basic.authorizerName", "basic") + .addCommonProperty("druid.auth.authorizers", "[\"basic\"]") + .addCommonProperty("druid.auth.authorizer.basic.type", "basic") + .addCommonProperty("druid.escalator.type", "basic") + .addCommonProperty("druid.escalator.internalClientPassword", "warlock") + .addCommonProperty("druid.escalator.internalClientUsername", "druid_system") + .addCommonProperty("druid.escalator.authorizerName", "basic"); + } +} diff --git a/integration-tests-ex/cases/src/test/java/org/apache/druid/testsEx/auth/ITSecurityBasicQuery.java b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/auth/BasicAuthMsqTest.java similarity index 53% rename from integration-tests-ex/cases/src/test/java/org/apache/druid/testsEx/auth/ITSecurityBasicQuery.java rename to embedded-tests/src/test/java/org/apache/druid/testing/embedded/auth/BasicAuthMsqTest.java index 3612faa9eb08..89a9be844468 100644 --- a/integration-tests-ex/cases/src/test/java/org/apache/druid/testsEx/auth/ITSecurityBasicQuery.java +++ b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/auth/BasicAuthMsqTest.java @@ -17,69 +17,140 @@ * under the License. */ -package org.apache.druid.testsEx.auth; +package org.apache.druid.testing.embedded.auth; import com.google.common.collect.ImmutableList; -import com.google.common.collect.ImmutableMap; -import com.google.inject.Inject; +import org.apache.calcite.sql.type.SqlTypeName; import org.apache.druid.common.utils.IdUtils; +import org.apache.druid.data.input.impl.JsonInputFormat; +import org.apache.druid.data.input.impl.LocalInputSource; +import org.apache.druid.error.ExceptionMatcher; +import org.apache.druid.indexing.common.task.Task; import org.apache.druid.java.util.common.StringUtils; -import org.apache.druid.java.util.http.client.response.StatusResponseHolder; +import org.apache.druid.metadata.DefaultPasswordProvider; +import org.apache.druid.msq.guice.IndexerMemoryManagementModule; +import org.apache.druid.msq.guice.MSQDurableStorageModule; +import org.apache.druid.msq.guice.MSQExternalDataSourceModule; +import org.apache.druid.msq.guice.MSQIndexingModule; +import org.apache.druid.msq.guice.MSQSqlModule; +import org.apache.druid.msq.guice.SqlTaskModule; +import org.apache.druid.msq.indexing.LegacyMSQSpec; import org.apache.druid.msq.indexing.MSQControllerTask; +import org.apache.druid.msq.indexing.MSQTuningConfig; +import org.apache.druid.msq.indexing.destination.ExportMSQDestination; +import org.apache.druid.msq.kernel.WorkerAssignmentStrategy; +import org.apache.druid.query.Druids; +import org.apache.druid.query.http.ClientSqlQuery; +import org.apache.druid.query.http.SqlTaskStatus; +import org.apache.druid.security.basic.BasicSecurityDruidModule; +import org.apache.druid.security.basic.authentication.BasicHTTPEscalator; +import org.apache.druid.segment.column.ColumnType; +import org.apache.druid.segment.column.RowSignature; import org.apache.druid.server.security.Action; import org.apache.druid.server.security.Resource; import org.apache.druid.server.security.ResourceAction; -import org.apache.druid.sql.http.SqlQuery; +import org.apache.druid.sql.calcite.external.ExternalDataSource; +import org.apache.druid.sql.calcite.planner.ColumnMapping; +import org.apache.druid.sql.calcite.planner.ColumnMappings; +import org.apache.druid.sql.http.ResultFormat; import org.apache.druid.storage.local.LocalFileExportStorageProvider; import org.apache.druid.storage.s3.output.S3ExportStorageProvider; -import org.apache.druid.testing.clients.CoordinatorResourceTestClient; -import org.apache.druid.testing.clients.OverlordResourceTestClient; -import org.apache.druid.testing.clients.SecurityClient; -import org.apache.druid.testing.utils.DataLoaderHelper; -import org.apache.druid.testing.utils.MsqTestQueryHelper; -import org.apache.druid.tests.indexer.AbstractIndexerTest; -import org.apache.druid.testsEx.categories.Security; -import org.apache.druid.testsEx.config.DruidTestRunner; -import org.jboss.netty.handler.codec.http.HttpResponseStatus; -import org.junit.After; -import org.junit.Assert; -import org.junit.Before; -import org.junit.Test; -import org.junit.experimental.categories.Category; -import org.junit.runner.RunWith; - -import java.io.IOException; +import org.apache.druid.testing.embedded.EmbeddedBroker; +import org.apache.druid.testing.embedded.EmbeddedCoordinator; +import org.apache.druid.testing.embedded.EmbeddedDruidCluster; +import org.apache.druid.testing.embedded.EmbeddedHistorical; +import org.apache.druid.testing.embedded.EmbeddedIndexer; +import org.apache.druid.testing.embedded.EmbeddedOverlord; +import org.apache.druid.testing.embedded.EmbeddedRouter; +import org.apache.druid.testing.embedded.EmbeddedServiceClient; +import org.apache.druid.testing.embedded.indexing.Resources; +import org.apache.druid.testing.embedded.junit5.EmbeddedClusterTestBase; +import org.apache.druid.testing.embedded.msq.MsqExportDirectory; +import org.hamcrest.MatcherAssert; +import org.junit.jupiter.api.AfterEach; +import org.junit.jupiter.api.Assertions; +import org.junit.jupiter.api.BeforeAll; +import org.junit.jupiter.api.BeforeEach; +import org.junit.jupiter.api.Test; + +import java.io.File; import java.util.List; -import java.util.concurrent.ExecutionException; +import java.util.Map; -@RunWith(DruidTestRunner.class) -@Category(Security.class) -public class ITSecurityBasicQuery +public class BasicAuthMsqTest extends EmbeddedClusterTestBase { - @Inject - private MsqTestQueryHelper msqHelper; - - @Inject - private DataLoaderHelper dataLoaderHelper; - - @Inject - private CoordinatorResourceTestClient coordinatorClient; - @Inject private SecurityClient securityClient; - @Inject - private OverlordResourceTestClient overlordResourceTestClient; public static final String USER_1 = "user1"; public static final String ROLE_1 = "role1"; public static final String USER_1_PASSWORD = "password1"; - private static final String EXPORT_TASK = "/indexer/export_task.json"; // Time in ms to sleep after updating role permissions in each test. This intends to give the // underlying test cluster enough time to sync permissions and be ready when test execution starts. - private static final int SYNC_SLEEP = 10000; + private static final int SYNC_SLEEP = 500; + + protected final EmbeddedBroker broker = new EmbeddedBroker(); + protected final EmbeddedIndexer indexer = new EmbeddedIndexer() + .setServerMemory(400_000_000) + .addProperty("druid.worker.capacity", "2"); + protected final EmbeddedOverlord overlord = new EmbeddedOverlord(); + protected final EmbeddedHistorical historical = new EmbeddedHistorical(); + protected final EmbeddedCoordinator coordinator = new EmbeddedCoordinator(); - @Before - public void setUp() throws IOException + private EmbeddedServiceClient userClient; + private final MsqExportDirectory exportDirectory = new MsqExportDirectory(); + + @Override + protected EmbeddedDruidCluster createCluster() + { + return EmbeddedDruidCluster + .withEmbeddedDerbyAndZookeeper() + .useLatchableEmitter() + .addResource(exportDirectory) + .addServer(coordinator) + .addServer(overlord) + .addServer(indexer) + .addServer(historical) + .addServer(broker) + .addServer(new EmbeddedRouter()) + .addExtensions( + BasicSecurityDruidModule.class, + MSQSqlModule.class, + MSQIndexingModule.class, + SqlTaskModule.class, + MSQDurableStorageModule.class, + MSQExternalDataSourceModule.class, + IndexerMemoryManagementModule.class + ) + .addCommonProperty("druid.auth.basic.common.pollingPeriod", "100") + .addCommonProperty("druid.auth.authenticatorChain", "[\"basic\"]") + .addCommonProperty("druid.auth.authenticator.basic.type", "basic") + .addCommonProperty("druid.auth.authenticator.basic.initialAdminPassword", "priest") + .addCommonProperty("druid.auth.authenticator.basic.initialInternalClientPassword", "warlock") + .addCommonProperty("druid.auth.authenticator.basic.authorizerName", "basic") + .addCommonProperty("druid.auth.authorizers", "[\"basic\"]") + .addCommonProperty("druid.auth.authorizer.basic.type", "basic") + .addCommonProperty("druid.escalator.type", "basic") + .addCommonProperty("druid.escalator.internalClientPassword", "warlock") + .addCommonProperty("druid.escalator.internalClientUsername", "druid_system") + .addCommonProperty("druid.escalator.authorizerName", "basic"); + } + + @BeforeAll + public void setupClient() + { + // Make a custom client for testing out auth for the test user + userClient = EmbeddedServiceClient.create( + cluster, + new BasicHTTPEscalator("basic", USER_1, new DefaultPasswordProvider(USER_1_PASSWORD)) + ); + + // Use the default set of clients for calling security APIs + securityClient = new SecurityClient(cluster.callApi().serviceClients()); + } + + @BeforeEach + public void setupRoles() { // Authentication setup securityClient.createAuthenticationUser(USER_1); @@ -91,8 +162,8 @@ public void setUp() throws IOException securityClient.assignUserToRole(USER_1, ROLE_1); } - @After - public void tearDown() throws Exception + @AfterEach + public void tearDownRoles() { securityClient.deleteAuthenticationUser(USER_1); securityClient.deleteAuthorizerUser(USER_1); @@ -105,8 +176,7 @@ public void testIngestionWithoutPermissions() throws Exception List permissions = ImmutableList.of(); securityClient.setPermissionsToRole(ROLE_1, permissions); - // Allow permissions sync across cluster to avoid flakes - Thread.sleep(SYNC_SLEEP); + waitForPermissionsToSync(); String queryLocal = StringUtils.format( @@ -138,23 +208,17 @@ public void testIngestionWithoutPermissions() throws Exception + " regionIsoCode\n" + "FROM TABLE(\n" + " EXTERN(\n" - + " '{\"type\":\"local\",\"files\":[\"/resources/data/batch_index/json/wikipedia_index_data1.json\"]}',\n" + + " '{\"type\":\"local\",\"files\":[\"%s\"]}',\n" + " '{\"type\":\"json\"}',\n" + " '[{\"type\":\"string\",\"name\":\"timestamp\"},{\"type\":\"string\",\"name\":\"isRobot\"},{\"type\":\"string\",\"name\":\"diffUrl\"},{\"type\":\"long\",\"name\":\"added\"},{\"type\":\"string\",\"name\":\"countryIsoCode\"},{\"type\":\"string\",\"name\":\"regionName\"},{\"type\":\"string\",\"name\":\"channel\"},{\"type\":\"string\",\"name\":\"flags\"},{\"type\":\"long\",\"name\":\"delta\"},{\"type\":\"string\",\"name\":\"isUnpatrolled\"},{\"type\":\"string\",\"name\":\"isNew\"},{\"type\":\"double\",\"name\":\"deltaBucket\"},{\"type\":\"string\",\"name\":\"isMinor\"},{\"type\":\"string\",\"name\":\"isAnonymous\"},{\"type\":\"long\",\"name\":\"deleted\"},{\"type\":\"string\",\"name\":\"cityName\"},{\"type\":\"long\",\"name\":\"metroCode\"},{\"type\":\"string\",\"name\":\"namespace\"},{\"type\":\"string\",\"name\":\"comment\"},{\"type\":\"string\",\"name\":\"page\"},{\"type\":\"long\",\"name\":\"commentLength\"},{\"type\":\"string\",\"name\":\"countryName\"},{\"type\":\"string\",\"name\":\"user\"},{\"type\":\"string\",\"name\":\"regionIsoCode\"}]'\n" + " )\n" + ")\n" + "PARTITIONED BY DAY\n", - "dst" + dataSource, + Resources.DataFile.tinyWiki1Json().getAbsolutePath() ); - // Submit the task and wait for the datasource to get loaded - StatusResponseHolder statusResponseHolder = msqHelper.submitMsqTask( - new SqlQuery(queryLocal, null, false, false, false, ImmutableMap.of(), ImmutableList.of()), - USER_1, - USER_1_PASSWORD - ); - - Assert.assertEquals(HttpResponseStatus.FORBIDDEN, statusResponseHolder.getStatus()); + verifySqlSubmitFailsWith403Forbidden(queryLocal); } @Test @@ -168,8 +232,7 @@ public void testIngestionWithPermissions() throws Exception ); securityClient.setPermissionsToRole(ROLE_1, permissions); - // Allow permissions sync across cluster to avoid flakes - Thread.sleep(SYNC_SLEEP); + waitForPermissionsToSync(); String queryLocal = StringUtils.format( @@ -201,27 +264,26 @@ public void testIngestionWithPermissions() throws Exception + " regionIsoCode\n" + "FROM TABLE(\n" + " EXTERN(\n" - + " '{\"type\":\"local\",\"files\":[\"/resources/data/batch_index/json/wikipedia_index_data1.json\"]}',\n" + + " '{\"type\":\"local\",\"files\":[\"%s\"]}',\n" + " '{\"type\":\"json\"}',\n" + " '[{\"type\":\"string\",\"name\":\"timestamp\"},{\"type\":\"string\",\"name\":\"isRobot\"},{\"type\":\"string\",\"name\":\"diffUrl\"},{\"type\":\"long\",\"name\":\"added\"},{\"type\":\"string\",\"name\":\"countryIsoCode\"},{\"type\":\"string\",\"name\":\"regionName\"},{\"type\":\"string\",\"name\":\"channel\"},{\"type\":\"string\",\"name\":\"flags\"},{\"type\":\"long\",\"name\":\"delta\"},{\"type\":\"string\",\"name\":\"isUnpatrolled\"},{\"type\":\"string\",\"name\":\"isNew\"},{\"type\":\"double\",\"name\":\"deltaBucket\"},{\"type\":\"string\",\"name\":\"isMinor\"},{\"type\":\"string\",\"name\":\"isAnonymous\"},{\"type\":\"long\",\"name\":\"deleted\"},{\"type\":\"string\",\"name\":\"cityName\"},{\"type\":\"long\",\"name\":\"metroCode\"},{\"type\":\"string\",\"name\":\"namespace\"},{\"type\":\"string\",\"name\":\"comment\"},{\"type\":\"string\",\"name\":\"page\"},{\"type\":\"long\",\"name\":\"commentLength\"},{\"type\":\"string\",\"name\":\"countryName\"},{\"type\":\"string\",\"name\":\"user\"},{\"type\":\"string\",\"name\":\"regionIsoCode\"}]'\n" + " )\n" + ")\n" + "PARTITIONED BY DAY\n", - "dst" + dataSource, + Resources.DataFile.tinyWiki1Json().getAbsolutePath() ); - // Submit the task and wait for the datasource to get loaded - StatusResponseHolder statusResponseHolder = msqHelper.submitMsqTask( - new SqlQuery(queryLocal, null, false, false, false, ImmutableMap.of(), ImmutableList.of()), - USER_1, - USER_1_PASSWORD + final SqlTaskStatus taskStatus = userClient.onAnyBroker( + b -> b.submitSqlTask( + new ClientSqlQuery(queryLocal, null, false, false, false, Map.of(), List.of()) + ) ); - - Assert.assertEquals(HttpResponseStatus.ACCEPTED, statusResponseHolder.getStatus()); + cluster.callApi().waitForTaskToSucceed(taskStatus.getTaskId(), overlord); } @Test - public void testExportWithoutPermissions() throws IOException, ExecutionException, InterruptedException + public void testExportWithoutPermissions() throws InterruptedException { // No external write permissions for s3 List permissions = ImmutableList.of( @@ -233,8 +295,7 @@ public void testExportWithoutPermissions() throws IOException, ExecutionExceptio ); securityClient.setPermissionsToRole(ROLE_1, permissions); - // Allow permissions sync across cluster to avoid flakes - Thread.sleep(SYNC_SLEEP); + waitForPermissionsToSync(); String exportQuery = StringUtils.format( @@ -243,25 +304,21 @@ public void testExportWithoutPermissions() throws IOException, ExecutionExceptio + "SELECT page, added, delta\n" + "FROM TABLE(\n" + " EXTERN(\n" - + " '{\"type\":\"local\",\"files\":[\"/resources/data/batch_index/json/wikipedia_index_data1.json\"]}',\n" + + " '{\"type\":\"local\",\"files\":[\"%s\"]}',\n" + " '{\"type\":\"json\"}',\n" + " '[{\"type\":\"string\",\"name\":\"timestamp\"},{\"type\":\"string\",\"name\":\"isRobot\"},{\"type\":\"string\",\"name\":\"diffUrl\"},{\"type\":\"long\",\"name\":\"added\"},{\"type\":\"string\",\"name\":\"countryIsoCode\"},{\"type\":\"string\",\"name\":\"regionName\"},{\"type\":\"string\",\"name\":\"channel\"},{\"type\":\"string\",\"name\":\"flags\"},{\"type\":\"long\",\"name\":\"delta\"},{\"type\":\"string\",\"name\":\"isUnpatrolled\"},{\"type\":\"string\",\"name\":\"isNew\"},{\"type\":\"double\",\"name\":\"deltaBucket\"},{\"type\":\"string\",\"name\":\"isMinor\"},{\"type\":\"string\",\"name\":\"isAnonymous\"},{\"type\":\"long\",\"name\":\"deleted\"},{\"type\":\"string\",\"name\":\"cityName\"},{\"type\":\"long\",\"name\":\"metroCode\"},{\"type\":\"string\",\"name\":\"namespace\"},{\"type\":\"string\",\"name\":\"comment\"},{\"type\":\"string\",\"name\":\"page\"},{\"type\":\"long\",\"name\":\"commentLength\"},{\"type\":\"string\",\"name\":\"countryName\"},{\"type\":\"string\",\"name\":\"user\"},{\"type\":\"string\",\"name\":\"regionIsoCode\"}]'\n" + " )\n" + ")\n", - LocalFileExportStorageProvider.TYPE_NAME, "/shared/export/" + LocalFileExportStorageProvider.TYPE_NAME, + cluster.getTestFolder().getOrCreateFolder("msq-export").getAbsolutePath(), + Resources.DataFile.tinyWiki1Json().getAbsolutePath() ); - StatusResponseHolder statusResponseHolder = msqHelper.submitMsqTask( - new SqlQuery(exportQuery, null, false, false, false, ImmutableMap.of(), ImmutableList.of()), - USER_1, - USER_1_PASSWORD - ); - - Assert.assertEquals(HttpResponseStatus.FORBIDDEN, statusResponseHolder.getStatus()); + verifySqlSubmitFailsWith403Forbidden(exportQuery); } @Test - public void testExportWithPermissions() throws IOException, ExecutionException, InterruptedException + public void testExportWithPermissions() throws InterruptedException { // No external write permissions for s3 List permissions = ImmutableList.of( @@ -273,8 +330,7 @@ public void testExportWithPermissions() throws IOException, ExecutionException, ); securityClient.setPermissionsToRole(ROLE_1, permissions); - // Allow permissions sync across cluster to avoid flakes - Thread.sleep(SYNC_SLEEP); + waitForPermissionsToSync(); String exportQuery = StringUtils.format( @@ -283,21 +339,22 @@ public void testExportWithPermissions() throws IOException, ExecutionException, + "SELECT page, added, delta\n" + "FROM TABLE(\n" + " EXTERN(\n" - + " '{\"type\":\"local\",\"files\":[\"/resources/data/batch_index/json/wikipedia_index_data1.json\"]}',\n" + + " '{\"type\":\"local\",\"files\":[\"%s\"]}',\n" + " '{\"type\":\"json\"}',\n" + " '[{\"type\":\"string\",\"name\":\"timestamp\"},{\"type\":\"string\",\"name\":\"isRobot\"},{\"type\":\"string\",\"name\":\"diffUrl\"},{\"type\":\"long\",\"name\":\"added\"},{\"type\":\"string\",\"name\":\"countryIsoCode\"},{\"type\":\"string\",\"name\":\"regionName\"},{\"type\":\"string\",\"name\":\"channel\"},{\"type\":\"string\",\"name\":\"flags\"},{\"type\":\"long\",\"name\":\"delta\"},{\"type\":\"string\",\"name\":\"isUnpatrolled\"},{\"type\":\"string\",\"name\":\"isNew\"},{\"type\":\"double\",\"name\":\"deltaBucket\"},{\"type\":\"string\",\"name\":\"isMinor\"},{\"type\":\"string\",\"name\":\"isAnonymous\"},{\"type\":\"long\",\"name\":\"deleted\"},{\"type\":\"string\",\"name\":\"cityName\"},{\"type\":\"long\",\"name\":\"metroCode\"},{\"type\":\"string\",\"name\":\"namespace\"},{\"type\":\"string\",\"name\":\"comment\"},{\"type\":\"string\",\"name\":\"page\"},{\"type\":\"long\",\"name\":\"commentLength\"},{\"type\":\"string\",\"name\":\"countryName\"},{\"type\":\"string\",\"name\":\"user\"},{\"type\":\"string\",\"name\":\"regionIsoCode\"}]'\n" + " )\n" + ")\n", - LocalFileExportStorageProvider.TYPE_NAME, "/shared/export/" + LocalFileExportStorageProvider.TYPE_NAME, + new File(exportDirectory.get(), dataSource).getAbsolutePath(), + Resources.DataFile.tinyWiki1Json() ); - StatusResponseHolder statusResponseHolder = msqHelper.submitMsqTask( - new SqlQuery(exportQuery, null, false, false, false, ImmutableMap.of(), ImmutableList.of()), - USER_1, - USER_1_PASSWORD + final SqlTaskStatus taskStatus = userClient.onAnyBroker( + b -> b.submitSqlTask( + new ClientSqlQuery(exportQuery, null, false, false, false, Map.of(), List.of()) + ) ); - - Assert.assertEquals(HttpResponseStatus.ACCEPTED, statusResponseHolder.getStatus()); + cluster.callApi().waitForTaskToSucceed(taskStatus.getTaskId(), overlord); } @Test @@ -313,12 +370,11 @@ public void testExportTaskSubmitOverlordWithPermission() throws Exception ); securityClient.setPermissionsToRole(ROLE_1, permissions); - // Allow permissions sync across cluster to avoid flakes - Thread.sleep(SYNC_SLEEP); + waitForPermissionsToSync(); - String task = createTaskString(); - StatusResponseHolder statusResponseHolder = overlordResourceTestClient.submitTaskAndReturnStatusWithAuth(task, USER_1, USER_1_PASSWORD); - Assert.assertEquals(HttpResponseStatus.OK, statusResponseHolder.getStatus()); + final String taskId = IdUtils.getRandomId(); + userClient.onLeaderOverlord(o -> o.runTask(taskId, createExportTask(taskId))); + cluster.callApi().waitForTaskToSucceed(taskId, overlord); } @Test @@ -334,18 +390,109 @@ public void testExportTaskSubmitOverlordWithoutPermission() throws Exception ); securityClient.setPermissionsToRole(ROLE_1, permissions); - // Allow permissions sync across cluster to avoid flakes - Thread.sleep(SYNC_SLEEP); + waitForPermissionsToSync(); - String task = createTaskString(); - StatusResponseHolder statusResponseHolder = overlordResourceTestClient.submitTaskAndReturnStatusWithAuth(task, USER_1, USER_1_PASSWORD); - Assert.assertEquals(HttpResponseStatus.FORBIDDEN, statusResponseHolder.getStatus()); + final String taskId = IdUtils.getRandomId(); + MatcherAssert.assertThat( + Assertions.assertThrows( + Exception.class, + () -> userClient.onLeaderOverlord(o -> o.runTask(taskId, createExportTask(taskId))) + ), + ExceptionMatcher.of(Exception.class).expectMessageContains("403 Forbidden") + ); } - private String createTaskString() throws Exception + private Task createExportTask(String taskId) { - String queryId = IdUtils.newTaskId(MSQControllerTask.TYPE, "external", null); - String template = AbstractIndexerTest.getResourceAsString(EXPORT_TASK); - return StringUtils.replace(template, "%%QUERY_ID%%", queryId); + final RowSignature rowSignature = RowSignature + .builder() + .add("timestamp", ColumnType.STRING) + .add("isRobot", ColumnType.STRING) + .add("diffUrl", ColumnType.STRING) + .add("added", ColumnType.LONG) + .add("countryIsoCode", ColumnType.STRING) + .add("regionName", ColumnType.STRING) + .add("channel", ColumnType.STRING) + .add("flags", ColumnType.STRING) + .add("delta", ColumnType.LONG) + .add("isUnpatrolled", ColumnType.STRING) + .add("isNew", ColumnType.STRING) + .add("deltaBucket", ColumnType.DOUBLE) + .add("isMinor", ColumnType.STRING) + .add("isAnonymous", ColumnType.STRING) + .add("deleted", ColumnType.LONG) + .add("cityName", ColumnType.STRING) + .add("metroCode", ColumnType.LONG) + .add("namespace", ColumnType.STRING) + .add("comment", ColumnType.STRING) + .add("page", ColumnType.STRING) + .add("commentLength", ColumnType.LONG) + .add("countryName", ColumnType.STRING) + .add("user", ColumnType.STRING) + .add("regionIsoCode", ColumnType.STRING) + .build(); + + return new MSQControllerTask( + taskId, + new LegacyMSQSpec( + new Druids.ScanQueryBuilder() + .columns("added", "delta", "page") + .dataSource( + new ExternalDataSource( + new LocalInputSource(null, null, List.of(Resources.DataFile.tinyWiki1Json()), null), + new JsonInputFormat(null, null, null, null, null), + rowSignature + ) + ) + .eternityInterval() + .context( + Map.of( + "scanSignature", + "[{\"name\":\"added\",\"type\":\"LONG\"},{\"name\":\"delta\",\"type\":\"LONG\"},{\"name\":\"page\",\"type\":\"STRING\"}]" + ) + ) + .build(), + new ColumnMappings( + List.of( + new ColumnMapping("page", "page"), + new ColumnMapping("added", "added"), + new ColumnMapping("delta", "delta") + ) + ), + new ExportMSQDestination( + new LocalFileExportStorageProvider(new File(exportDirectory.get(), dataSource).getAbsolutePath()), + ResultFormat.CSV + ), + WorkerAssignmentStrategy.MAX, + MSQTuningConfig.defaultConfig() + ), + null, + null, + null, + List.of(SqlTypeName.VARCHAR, SqlTypeName.BIGINT, SqlTypeName.BIGINT), + List.of(ColumnType.STRING, ColumnType.LONG, ColumnType.LONG), + null, + null + ); + } + + private void waitForPermissionsToSync() throws InterruptedException + { + Thread.sleep(SYNC_SLEEP); + } + + private void verifySqlSubmitFailsWith403Forbidden(String sql) + { + MatcherAssert.assertThat( + Assertions.assertThrows( + Exception.class, + () -> userClient.onAnyBroker( + b -> b.submitSqlTask( + new ClientSqlQuery(sql, null, false, false, false, Map.of(), List.of()) + ) + ) + ), + ExceptionMatcher.of(Exception.class).expectMessageContains("403 Forbidden") + ); } } diff --git a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/auth/SecurityClient.java b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/auth/SecurityClient.java new file mode 100644 index 000000000000..1c9da7210c84 --- /dev/null +++ b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/auth/SecurityClient.java @@ -0,0 +1,177 @@ +/* + * 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.testing.embedded.auth; + +import com.fasterxml.jackson.databind.ObjectMapper; +import org.apache.druid.java.util.common.StringUtils; +import org.apache.druid.rpc.RequestBuilder; +import org.apache.druid.server.security.ResourceAction; +import org.apache.druid.testing.embedded.EmbeddedServiceClient; +import org.jboss.netty.handler.codec.http.HttpMethod; + +import java.util.List; +import java.util.Map; +import java.util.function.Function; + +public class SecurityClient +{ + private final String coordinator; + private final EmbeddedServiceClient clients; + + SecurityClient(EmbeddedServiceClient clients) + { + this.clients = clients; + this.coordinator = "/druid/coordinator/v1"; + } + + public void createAuthenticationUser(String username) + { + final RequestBuilder request = new RequestBuilder( + HttpMethod.POST, + StringUtils.format( + "%s/users/%s", + getAuthenticatorURL(), + StringUtils.urlEncode(username) + ) + ); + sendRequest(mapper -> request); + } + + public void deleteAuthenticationUser(String username) + { + final RequestBuilder request = new RequestBuilder( + HttpMethod.DELETE, + StringUtils.format( + "%s/users/%s", + getAuthenticatorURL(), + StringUtils.urlEncode(username) + ) + ); + sendRequest(mapper -> request); + } + + public void setUserPassword(String username, String password) + { + final RequestBuilder request = new RequestBuilder( + HttpMethod.POST, + StringUtils.format( + "%s/users/%s/credentials", + getAuthenticatorURL(), + StringUtils.urlEncode(username) + ) + ); + + sendRequest(mapper -> request.jsonContent(mapper, Map.of("password", password))); + } + + public void createAuthorizerUser(String username) + { + final RequestBuilder request = new RequestBuilder( + HttpMethod.POST, + StringUtils.format( + "%s/users/%s", + getAuthorizerURL(), + StringUtils.urlEncode(username) + ) + ); + sendRequest(mapper -> request); + } + + public void deleteAuthorizerUser(String username) + { + final RequestBuilder request = new RequestBuilder( + HttpMethod.DELETE, + StringUtils.format( + "%s/users/%s", + getAuthorizerURL(), + StringUtils.urlEncode(username) + ) + ); + sendRequest(mapper -> request); + } + + public void createAuthorizerRole(String role) + { + final RequestBuilder request = new RequestBuilder( + HttpMethod.POST, + StringUtils.format( + "%s/roles/%s", + getAuthorizerURL(), + StringUtils.urlEncode(role) + ) + ); + sendRequest(mapper -> request); + } + + public void deleteAuthorizerRole(String role) + { + final RequestBuilder request = new RequestBuilder( + HttpMethod.DELETE, + StringUtils.format( + "%s/roles/%s", + getAuthorizerURL(), + StringUtils.urlEncode(role) + ) + ); + sendRequest(mapper -> request); + } + + public void assignUserToRole(String user, String role) + { + final RequestBuilder request = new RequestBuilder( + HttpMethod.POST, + StringUtils.format( + "%s/users/%s/roles/%s", + getAuthorizerURL(), + StringUtils.urlEncode(user), + StringUtils.urlEncode(role) + ) + ); + sendRequest(mapper -> request); + } + + public void setPermissionsToRole(String role, List permissions) + { + final RequestBuilder request = new RequestBuilder( + HttpMethod.POST, + StringUtils.format( + "%s/roles/%s/permissions/", + getAuthorizerURL(), + StringUtils.urlEncode(role) + ) + ); + sendRequest(mapper -> request.jsonContent(mapper, permissions)); + } + + private void sendRequest(Function request) + { + clients.onLeaderCoordinator(request, null); + } + + private String getAuthenticatorURL() + { + return "/druid-ext/basic-security/authentication/db/basic"; + } + + private String getAuthorizerURL() + { + return "/druid-ext/basic-security/authorization/db/basic"; + } +} diff --git a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/compact/AutoCompactionTest.java b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/compact/AutoCompactionTest.java index c867305e7221..1c2f7fca78b2 100644 --- a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/compact/AutoCompactionTest.java +++ b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/compact/AutoCompactionTest.java @@ -91,6 +91,7 @@ import org.joda.time.Period; import org.joda.time.chrono.ISOChronology; import org.junit.jupiter.api.Assertions; +import org.junit.jupiter.api.BeforeAll; import org.junit.jupiter.api.BeforeEach; import org.junit.jupiter.api.Test; import org.junit.jupiter.params.ParameterizedTest; @@ -213,11 +214,16 @@ protected EmbeddedDruidCluster createCluster() .addServer(new EmbeddedRouter()); } - protected final CompactionResourceTestClient compactionResource = - new CompactionResourceTestClient(coordinator, overlord); + protected CompactionResourceTestClient compactionResource; private String fullDatasourceName; + @BeforeAll + public void setupClient() + { + this.compactionResource = new CompactionResourceTestClient(cluster); + } + @BeforeEach public void resetCompactionTaskSlots() throws Exception { diff --git a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/compact/AutoCompactionUpgradeTest.java b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/compact/AutoCompactionUpgradeTest.java index 22155c6745e6..708ad28607e2 100644 --- a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/compact/AutoCompactionUpgradeTest.java +++ b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/compact/AutoCompactionUpgradeTest.java @@ -37,6 +37,7 @@ import org.apache.druid.testing.embedded.junit5.EmbeddedClusterTestBase; import org.joda.time.Period; import org.junit.jupiter.api.Assertions; +import org.junit.jupiter.api.BeforeAll; import org.junit.jupiter.api.Test; import java.nio.charset.StandardCharsets; @@ -47,8 +48,7 @@ public class AutoCompactionUpgradeTest extends EmbeddedClusterTestBase private final EmbeddedCoordinator coordinator = new EmbeddedCoordinator() .addProperty("druid.manager.segments.useIncrementalCache", "always"); - protected CompactionResourceTestClient compactionResource = - new CompactionResourceTestClient(coordinator, overlord); + protected CompactionResourceTestClient compactionResource; @Override protected EmbeddedDruidCluster createCluster() @@ -57,6 +57,12 @@ protected EmbeddedDruidCluster createCluster() .addServer(coordinator); } + @BeforeAll + public void setupClient() + { + compactionResource = new CompactionResourceTestClient(cluster); + } + @Test public void test_configCanBeUpdated_afterVersionUpgrades() throws Exception { diff --git a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/compact/CompactionResourceTestClient.java b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/compact/CompactionResourceTestClient.java index aafcf14a2bb9..359966a2f28f 100644 --- a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/compact/CompactionResourceTestClient.java +++ b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/compact/CompactionResourceTestClient.java @@ -20,32 +20,24 @@ package org.apache.druid.testing.embedded.compact; import com.fasterxml.jackson.core.type.TypeReference; -import com.fasterxml.jackson.databind.ObjectMapper; import org.apache.druid.indexing.overlord.http.CompactionConfigsResponse; -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.java.util.http.client.HttpClient; -import org.apache.druid.java.util.http.client.Request; -import org.apache.druid.java.util.http.client.response.StatusResponseHandler; -import org.apache.druid.java.util.http.client.response.StatusResponseHolder; -import org.apache.druid.query.aggregation.datasketches.hll.HllSketchModule; -import org.apache.druid.query.aggregation.datasketches.quantiles.DoublesSketchModule; -import org.apache.druid.query.aggregation.datasketches.theta.SketchModule; -import org.apache.druid.segment.TestHelper; +import org.apache.druid.rpc.RequestBuilder; +import org.apache.druid.rpc.indexing.OverlordClient; import org.apache.druid.server.compaction.CompactionSimulateResult; import org.apache.druid.server.compaction.CompactionStatusResponse; import org.apache.druid.server.coordinator.AutoCompactionSnapshot; import org.apache.druid.server.coordinator.ClusterCompactionConfig; import org.apache.druid.server.coordinator.DataSourceCompactionConfig; -import org.apache.druid.testing.embedded.EmbeddedCoordinator; -import org.apache.druid.testing.embedded.EmbeddedOverlord; +import org.apache.druid.server.http.ServletResourceUtils; +import org.apache.druid.testing.embedded.EmbeddedDruidCluster; +import org.apache.druid.testing.embedded.EmbeddedServiceClient; import org.jboss.netty.handler.codec.http.HttpMethod; -import org.jboss.netty.handler.codec.http.HttpResponseStatus; -import java.net.URL; import java.util.List; import java.util.Map; +import java.util.Objects; /** * The methods in this class should eventually be updated to use @@ -55,133 +47,72 @@ public class CompactionResourceTestClient { private static final Logger log = new Logger(CompactionResourceTestClient.class); - private final ObjectMapper jsonMapper; - private final EmbeddedOverlord overlord; - private final EmbeddedCoordinator coordinator; - private final StatusResponseHandler responseHandler; + private final EmbeddedServiceClient clients; - CompactionResourceTestClient( - EmbeddedCoordinator coordinator, - EmbeddedOverlord overlord - ) + CompactionResourceTestClient(EmbeddedDruidCluster cluster) { - this.jsonMapper = TestHelper.JSON_MAPPER - .registerModules(new SketchModule().getJacksonModules()) - .registerModules(new HllSketchModule().getJacksonModules()) - .registerModules(new DoublesSketchModule().getJacksonModules()); - this.overlord = overlord; - this.coordinator = coordinator; - this.responseHandler = StatusResponseHandler.getInstance(); - } - - private HttpClient httpClient() - { - return overlord.bindings().escalatedHttpClient(); + this.clients = cluster.callApi().serviceClients(); } private String getCoordinatorURL() { - return StringUtils.format( - "http://%s:%s/druid/coordinator/v1/", - coordinator.bindings().selfNode().getHost(), - coordinator.bindings().selfNode().getPlaintextPort() - ); + return "/druid/coordinator/v1"; } private String getOverlordURL() { - return StringUtils.format( - "http://%s:%s/druid/indexer/v1", - overlord.bindings().selfNode().getHost(), - overlord.bindings().selfNode().getPlaintextPort() - ); + return "/druid/indexer/v1"; } - public void submitCompactionConfig(final DataSourceCompactionConfig dataSourceCompactionConfig) throws Exception + public void submitCompactionConfig(final DataSourceCompactionConfig dataSourceCompactionConfig) { final String dataSource = dataSourceCompactionConfig.getDataSource(); String url = StringUtils.format( "%s/compaction/config/datasources/%s", getOverlordURL(), StringUtils.urlEncode(dataSource) ); - StatusResponseHolder response = httpClient().go( - new Request(HttpMethod.POST, new URL(url)).setContent( - "application/json", - jsonMapper.writeValueAsBytes(dataSourceCompactionConfig) + clients.onLeaderOverlord( + mapper -> new RequestBuilder(HttpMethod.POST, url).jsonContent( + mapper, + dataSourceCompactionConfig ), - responseHandler - ).get(); - - if (!response.getStatus().equals(HttpResponseStatus.OK)) { - throw new ISE( - "Error while submiting compaction config status[%s] content[%s]", - response.getStatus(), - response.getContent() - ); - } - log.info( - "Submitted compaction config for datasource[%s] with response[%s]", - dataSource, response.getContent() + null ); + log.info("Submitted compaction config for datasource[%s]", dataSource); } - public void deleteDataSourceCompactionConfig(final String dataSource) throws Exception + public void deleteDataSourceCompactionConfig(final String dataSource) { String url = StringUtils.format( "%s/compaction/config/datasources/%s", getOverlordURL(), StringUtils.urlEncode(dataSource) ); - StatusResponseHolder response = httpClient().go(new Request(HttpMethod.DELETE, new URL(url)), responseHandler).get(); - - if (!response.getStatus().equals(HttpResponseStatus.OK)) { - throw new ISE( - "Error while deleting compaction config status[%s] content[%s]", - response.getStatus(), - response.getContent() - ); - } + clients.onLeaderOverlord(mapper -> new RequestBuilder(HttpMethod.DELETE, url), null); } - public List getAllCompactionConfigs() throws Exception + public List getAllCompactionConfigs() { String url = StringUtils.format("%s/compaction/config/datasources", getOverlordURL()); - StatusResponseHolder response = httpClient().go( - new Request(HttpMethod.GET, new URL(url)), responseHandler - ).get(); - if (!response.getStatus().equals(HttpResponseStatus.OK)) { - throw new ISE( - "Error while getting compaction config status[%s] content[%s]", - response.getStatus(), - response.getContent() - ); - } - final CompactionConfigsResponse payload = jsonMapper.readValue( - response.getContent(), + final CompactionConfigsResponse payload = clients.onLeaderOverlord( + mapper -> new RequestBuilder(HttpMethod.GET, url), new TypeReference<>() {} ); - return payload.getCompactionConfigs(); + return Objects.requireNonNull(payload).getCompactionConfigs(); } - public DataSourceCompactionConfig getDataSourceCompactionConfig(String dataSource) throws Exception + public DataSourceCompactionConfig getDataSourceCompactionConfig(String dataSource) { String url = StringUtils.format( "%s/compaction/config/datasources/%s", getOverlordURL(), StringUtils.urlEncode(dataSource) ); - StatusResponseHolder response = httpClient().go( - new Request(HttpMethod.GET, new URL(url)), responseHandler - ).get(); - if (!response.getStatus().equals(HttpResponseStatus.OK)) { - throw new ISE( - "Error while getting compaction config status[%s] content[%s]", - response.getStatus(), - response.getContent() - ); - } - return jsonMapper.readValue(response.getContent(), new TypeReference<>() {}); + return clients.onLeaderOverlord( + mapper -> new RequestBuilder(HttpMethod.GET, url), + new TypeReference<>() {} + ); } - public void forceTriggerAutoCompaction() throws Exception + public void forceTriggerAutoCompaction() { // Perform a dummy update of task slots to force the coordinator to refresh its compaction config final ClusterCompactionConfig clusterConfig = getClusterConfig(); @@ -199,53 +130,18 @@ public void forceTriggerAutoCompaction() throws Exception simulateResult.getCompactionStates() ); - String url = StringUtils.format("%scompaction/compact", getCoordinatorURL()); - StatusResponseHolder response = httpClient().go(new Request(HttpMethod.POST, new URL(url)), responseHandler).get(); - if (!response.getStatus().equals(HttpResponseStatus.OK)) { - throw new ISE( - "Error while force trigger auto compaction status[%s] content[%s]", - response.getStatus(), - response.getContent() - ); - } + String url = StringUtils.format("%s/compaction/compact", getCoordinatorURL()); + clients.onLeaderCoordinator(mapper -> new RequestBuilder(HttpMethod.POST, url), null); } - public void updateClusterConfig(ClusterCompactionConfig config) throws Exception + public void updateClusterConfig(ClusterCompactionConfig config) { - final String url = StringUtils.format( - "%s/compaction/config/cluster", - getOverlordURL() - ); - StatusResponseHolder response = httpClient().go( - new Request(HttpMethod.POST, new URL(url)).setContent( - "application/json", - jsonMapper.writeValueAsBytes(config) - ), - responseHandler - ).get(); - if (!response.getStatus().equals(HttpResponseStatus.OK)) { - throw new ISE( - "Error while updating cluster compaction config, status[%s], content[%s]", - response.getStatus(), - response.getContent() - ); - } + clients.onLeaderOverlord(o -> o.updateClusterCompactionConfig(config)); } - public ClusterCompactionConfig getClusterConfig() throws Exception + public ClusterCompactionConfig getClusterConfig() { - String url = StringUtils.format("%s/compaction/config/cluster", getOverlordURL()); - StatusResponseHolder response = httpClient().go( - new Request(HttpMethod.GET, new URL(url)), responseHandler - ).get(); - if (!response.getStatus().equals(HttpResponseStatus.OK)) { - throw new ISE( - "Error while getting compaction config status[%s] content[%s]", - response.getStatus(), - response.getContent() - ); - } - return jsonMapper.readValue(response.getContent(), new TypeReference<>() {}); + return clients.onLeaderOverlord(OverlordClient::getClusterCompactionConfig); } /** @@ -253,78 +149,58 @@ public ClusterCompactionConfig getClusterConfig() throws Exception * For all other purposes, use {@link #updateClusterConfig}. */ @Deprecated - private void updateCompactionTaskSlot(Double compactionTaskSlotRatio, Integer maxCompactionTaskSlots) throws Exception + private void updateCompactionTaskSlot(Double compactionTaskSlotRatio, Integer maxCompactionTaskSlots) { final String url = StringUtils.format( - "%sconfig/compaction/taskslots?ratio=%s&max=%s", + "%s/config/compaction/taskslots?ratio=%s&max=%s", getCoordinatorURL(), StringUtils.urlEncode(compactionTaskSlotRatio.toString()), StringUtils.urlEncode(maxCompactionTaskSlots.toString()) ); - StatusResponseHolder response = httpClient().go(new Request(HttpMethod.POST, new URL(url)), responseHandler).get(); - if (!response.getStatus().equals(HttpResponseStatus.OK)) { - throw new ISE( - "Error while updating compaction task slot status[%s] content[%s]", - response.getStatus(), - response.getContent() - ); - } + clients.onLeaderCoordinator(mapper -> new RequestBuilder(HttpMethod.POST, url), null); } - public Map getCompactionProgress(String dataSource) throws Exception + public Map getCompactionProgress(String dataSource) { - String url = StringUtils.format("%scompaction/progress?dataSource=%s", getCoordinatorURL(), StringUtils.urlEncode(dataSource)); - StatusResponseHolder response = httpClient().go( - new Request(HttpMethod.GET, new URL(url)), responseHandler - ).get(); - if (!response.getStatus().equals(HttpResponseStatus.OK)) { - throw new ISE( - "Error while getting compaction progress status[%s] content[%s]", - response.getStatus(), - response.getContent() - ); - } - return jsonMapper.readValue(response.getContent(), new TypeReference<>() {}); + String url = StringUtils.format( + "%s/compaction/progress?dataSource=%s", + getCoordinatorURL(), + StringUtils.urlEncode(dataSource) + ); + return clients.onLeaderCoordinator( + mapper -> new RequestBuilder(HttpMethod.GET, url), + new TypeReference<>() {} + ); } - public AutoCompactionSnapshot getCompactionStatus(String dataSource) throws Exception + public AutoCompactionSnapshot getCompactionStatus(String dataSource) { - String url = StringUtils.format("%scompaction/status?dataSource=%s", getCoordinatorURL(), StringUtils.urlEncode(dataSource)); - StatusResponseHolder response = httpClient().go( - new Request(HttpMethod.GET, new URL(url)), responseHandler - ).get(); - if (response.getStatus().equals(HttpResponseStatus.NOT_FOUND)) { - return null; - } else if (!response.getStatus().equals(HttpResponseStatus.OK)) { - throw new ISE( - "Error while getting compaction status status[%s] content[%s]", - response.getStatus(), - response.getContent() + String url = StringUtils.format( + "%s/compaction/status?dataSource=%s", + getCoordinatorURL(), + StringUtils.urlEncode(dataSource) + ); + + try { + final CompactionStatusResponse latestSnapshots = clients.onLeaderCoordinator( + mapper -> new RequestBuilder(HttpMethod.GET, url), + new TypeReference<>() {} ); + return Objects.requireNonNull(latestSnapshots).getLatestStatus().get(0); + } + catch (Exception e) { + return ServletResourceUtils.getDefaultValueIfCauseIs404ElseThrow(e, () -> null); } - final CompactionStatusResponse latestSnapshots = jsonMapper.readValue(response.getContent(), new TypeReference<>() {}); - return latestSnapshots.getLatestStatus().get(0); } - public CompactionSimulateResult simulateRunOnCoordinator() throws Exception + public CompactionSimulateResult simulateRunOnCoordinator() { final ClusterCompactionConfig clusterConfig = getClusterConfig(); - final String url = StringUtils.format("%scompaction/simulate", getCoordinatorURL()); - StatusResponseHolder response = httpClient().go( - new Request(HttpMethod.POST, new URL(url)).setContent( - "application/json", - jsonMapper.writeValueAsBytes(clusterConfig) - ), - responseHandler - ).get(); - if (!response.getStatus().equals(HttpResponseStatus.OK)) { - throw new ISE( - "Error while running simulation on Coordinator: status[%s], content[%s]", - response.getStatus(), response.getContent() - ); - } - - return jsonMapper.readValue(response.getContent(), new TypeReference<>() {}); + final String url = StringUtils.format("%s/compaction/simulate", getCoordinatorURL()); + return clients.onLeaderCoordinator( + mapper -> new RequestBuilder(HttpMethod.POST, url).jsonContent(mapper, clusterConfig), + new TypeReference<>() {} + ); } } diff --git a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/compact/CompactionSparseColumnTest.java b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/compact/CompactionSparseColumnTest.java index f819095b6496..2777d244609d 100644 --- a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/compact/CompactionSparseColumnTest.java +++ b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/compact/CompactionSparseColumnTest.java @@ -22,7 +22,6 @@ import com.fasterxml.jackson.core.type.TypeReference; import com.google.common.collect.ImmutableList; import com.google.common.collect.ImmutableMap; -import org.apache.druid.common.guava.FutureUtils; import org.apache.druid.indexer.partitions.DynamicPartitionsSpec; import org.apache.druid.indexer.partitions.HashedPartitionsSpec; import org.apache.druid.indexing.common.task.TaskBuilder; @@ -238,8 +237,7 @@ private void verifyQueryResult( */ private String getScanEvents(ScanQuery scanQuery) { - final String resultAsJson = - FutureUtils.getUnchecked(cluster.anyBroker().submitNativeQuery(scanQuery), true); + final String resultAsJson = cluster.callApi().onAnyBroker(b -> b.submitNativeQuery(scanQuery)); final List> resultList = JacksonUtils.readValue( TestHelper.JSON_MAPPER, resultAsJson.getBytes(StandardCharsets.UTF_8), diff --git a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/compact/CompactionTaskTest.java b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/compact/CompactionTaskTest.java index a9ad05ca2b53..99eb428cf9c7 100644 --- a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/compact/CompactionTaskTest.java +++ b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/compact/CompactionTaskTest.java @@ -21,7 +21,6 @@ import com.fasterxml.jackson.core.type.TypeReference; import org.apache.druid.client.indexing.ClientCompactionTaskGranularitySpec; -import org.apache.druid.common.guava.FutureUtils; import org.apache.druid.indexer.partitions.HashedPartitionsSpec; import org.apache.druid.indexer.report.IngestionStatsAndErrors; import org.apache.druid.indexer.report.IngestionStatsAndErrorsTaskReport; @@ -321,10 +320,7 @@ private void verifySegmentsHaveQueryGranularity(String expectedQueryGranularity, .intervals("2013-08-31/2013-09-02") .build(); - final String resultAsJson = FutureUtils.getUnchecked( - cluster.anyBroker().submitNativeQuery(query), - true - ); + final String resultAsJson = cluster.callApi().onAnyBroker(b -> b.submitNativeQuery(query)); // Trim the result so that it contains only the `queryGranularity` fields final List> resultList = JacksonUtils.readValue( diff --git a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/indexing/IndexParallelTaskTest.java b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/indexing/IndexParallelTaskTest.java index e97c9b977473..8127e33027fc 100644 --- a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/indexing/IndexParallelTaskTest.java +++ b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/indexing/IndexParallelTaskTest.java @@ -100,7 +100,7 @@ public void test_segmentAvailabilityIsConfirmed_whenTaskWaits5secondsForHandoff( .dataSource(dataSource) .timestampColumn("timestamp") .jsonInputFormat() - .localInputSourceWithFiles(Resources.DataFile.TINY_WIKI_1_JSON) + .localInputSourceWithFiles(Resources.DataFile.tinyWiki1Json()) .dimensions() .tuningConfig( t -> t.withAwaitSegmentAvailabilityTimeoutMillis(segmentAvailabilityTimeoutMillis) @@ -135,9 +135,9 @@ public void test_runIndexTask_andReindexIntoAnotherDatasource(PartitionsSpec par .timestampColumn("timestamp") .jsonInputFormat() .localInputSourceWithFiles( - Resources.DataFile.TINY_WIKI_1_JSON, - Resources.DataFile.TINY_WIKI_2_JSON, - Resources.DataFile.TINY_WIKI_3_JSON + Resources.DataFile.tinyWiki1Json(), + Resources.DataFile.tinyWiki2Json(), + Resources.DataFile.tinyWiki3Json() ) .segmentGranularity("DAY") .dimensions("namespace", "page", "language") diff --git a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/indexing/MoreResources.java b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/indexing/MoreResources.java index 539ec5d0680b..57a18694e10d 100644 --- a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/indexing/MoreResources.java +++ b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/indexing/MoreResources.java @@ -44,9 +44,9 @@ public static class Task .ofTypeIndex() .jsonInputFormat() .localInputSourceWithFiles( - Resources.DataFile.TINY_WIKI_1_JSON, - Resources.DataFile.TINY_WIKI_2_JSON, - Resources.DataFile.TINY_WIKI_3_JSON + Resources.DataFile.tinyWiki1Json(), + Resources.DataFile.tinyWiki2Json(), + Resources.DataFile.tinyWiki3Json() ) .timestampColumn("timestamp") .dimensions( diff --git a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/msq/BaseRealtimeQueryTest.java b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/msq/BaseRealtimeQueryTest.java index 42baf6a86623..bce1362254f1 100644 --- a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/msq/BaseRealtimeQueryTest.java +++ b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/msq/BaseRealtimeQueryTest.java @@ -134,9 +134,10 @@ void tearDownEach() throws ExecutionException, InterruptedException, IOException Assertions.assertEquals(Map.of("id", dataSource), terminateSupervisorResult); // Cancel all running tasks, so we don't need to wait for them to hand off their segments. - try (final CloseableIterator it = cluster.leaderOverlord().taskStatuses(null, null, null).get()) { + try (final CloseableIterator it + = cluster.callApi().onLeaderOverlord(o -> o.taskStatuses(null, null, null))) { while (it.hasNext()) { - cluster.leaderOverlord().cancelTask(it.next().getId()); + cluster.callApi().onLeaderOverlord(o -> o.cancelTask(it.next().getId())); } } diff --git a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/msq/EmbeddedMSQApis.java b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/msq/EmbeddedMSQApis.java index 4c002039f4b1..71967443d40c 100644 --- a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/msq/EmbeddedMSQApis.java +++ b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/msq/EmbeddedMSQApis.java @@ -19,7 +19,6 @@ package org.apache.druid.testing.embedded.msq; -import org.apache.druid.common.guava.FutureUtils; import org.apache.druid.error.DruidException; import org.apache.druid.indexer.TaskState; import org.apache.druid.indexer.report.TaskReport; @@ -62,8 +61,8 @@ public EmbeddedMSQApis(EmbeddedDruidCluster cluster, EmbeddedOverlord overlord) */ public String runDartSql(String sql, Object... args) { - return FutureUtils.getUnchecked( - cluster.anyBroker().submitSqlQuery( + return cluster.callApi().onAnyBroker( + b -> b.submitSqlQuery( new ClientSqlQuery( StringUtils.format(sql, args), ResultFormat.CSV.name(), @@ -73,8 +72,7 @@ public String runDartSql(String sql, Object... args) Map.of(QueryContexts.ENGINE, DartSqlEngine.NAME), null ) - ), - true + ) ).trim(); } @@ -87,8 +85,8 @@ public String runDartSql(String sql, Object... args) public SqlTaskStatus submitTaskSql(String sql, Object... args) { final SqlTaskStatus taskStatus = - FutureUtils.getUnchecked( - cluster.anyBroker().submitSqlTask( + cluster.callApi().onAnyBroker( + b -> b.submitSqlTask( new ClientSqlQuery( StringUtils.format(sql, args), ResultFormat.CSV.name(), @@ -98,8 +96,7 @@ public SqlTaskStatus submitTaskSql(String sql, Object... args) null, null ) - ), - true + ) ); if (taskStatus.getState() != TaskState.RUNNING) { @@ -125,9 +122,8 @@ public MSQTaskReportPayload runTaskSqlAndGetReport(String sql, Object... args) cluster.callApi().waitForTaskToSucceed(taskStatus.getTaskId(), overlord); - final TaskReport.ReportMap taskReport = FutureUtils.getUnchecked( - cluster.leaderOverlord().taskReportAsMap(taskStatus.getTaskId()), - true + final TaskReport.ReportMap taskReport = cluster.callApi().onLeaderOverlord( + o -> o.taskReportAsMap(taskStatus.getTaskId()) ); final Optional report = taskReport.findReport(MSQTaskReport.REPORT_KEY); diff --git a/services/src/test/java/org/apache/druid/testing/embedded/ClusterReferencesProvider.java b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/msq/MsqExportDirectory.java similarity index 52% rename from services/src/test/java/org/apache/druid/testing/embedded/ClusterReferencesProvider.java rename to embedded-tests/src/test/java/org/apache/druid/testing/embedded/msq/MsqExportDirectory.java index 70f96aeb2a75..43fd18335c9e 100644 --- a/services/src/test/java/org/apache/druid/testing/embedded/ClusterReferencesProvider.java +++ b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/msq/MsqExportDirectory.java @@ -17,31 +17,41 @@ * under the License. */ -package org.apache.druid.testing.embedded; +package org.apache.druid.testing.embedded.msq; -import org.apache.druid.client.broker.BrokerClient; -import org.apache.druid.client.coordinator.CoordinatorClient; -import org.apache.druid.rpc.indexing.OverlordClient; +import org.apache.druid.testing.embedded.EmbeddedDruidCluster; +import org.apache.druid.testing.embedded.EmbeddedResource; + +import java.io.File; /** - * Provides a handle to the various service clients being used by an - * {@link EmbeddedDruidCluster}. + * Embedded resource to set MSQ export directory. */ -public interface ClusterReferencesProvider +public class MsqExportDirectory implements EmbeddedResource { - /** - * Client to make API calls to the leader Coordinator in the cluster. - */ - CoordinatorClient leaderCoordinator(); + private File directory; + + @Override + public void start() + { + // Do nothing + } - /** - * Client to make API calls to the leader Overlord in the cluster. - */ - OverlordClient leaderOverlord(); + @Override + public void stop() + { + // Do nothing + } - /** - * Client to submit queries to any Broker in the cluster. - */ - BrokerClient anyBroker(); + @Override + public void onStarted(EmbeddedDruidCluster cluster) + { + directory = cluster.getTestFolder().getOrCreateFolder("msq-export"); + cluster.addCommonProperty("druid.export.storage.baseDir", directory.getAbsolutePath()); + } + public File get() + { + return directory; + } } diff --git a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/server/OverlordClientTest.java b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/server/OverlordClientTest.java index 25261553190a..36de4c072a7f 100644 --- a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/server/OverlordClientTest.java +++ b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/server/OverlordClientTest.java @@ -21,10 +21,7 @@ import com.amazonaws.util.Throwables; import com.google.common.collect.ImmutableList; -import com.google.common.util.concurrent.FutureCallback; -import com.google.common.util.concurrent.Futures; import com.google.common.util.concurrent.ListenableFuture; -import com.google.common.util.concurrent.MoreExecutors; import org.apache.druid.client.indexing.IndexingTotalWorkerCapacityInfo; import org.apache.druid.client.indexing.IndexingWorkerInfo; import org.apache.druid.common.utils.IdUtils; @@ -58,8 +55,7 @@ import java.util.List; import java.util.Map; import java.util.Set; -import java.util.concurrent.CountDownLatch; -import java.util.concurrent.atomic.AtomicReference; +import java.util.function.Function; import java.util.stream.Collectors; /** @@ -114,7 +110,7 @@ public void test_runKillTask() public void test_cancelTask_fails_forUnknownTaskId() { verifyApiFailsWith( - cluster.leaderOverlord().cancelTask(UNKNOWN_TASK_ID), + o -> o.cancelTask(UNKNOWN_TASK_ID), UNKNOWN_TASK_ERROR ); } @@ -182,7 +178,7 @@ public void test_taskStatuses_forCompleteTasks() public void test_taskStatuses_byIds_returnsEmpty_forUnknownTaskIds() { Map result = cluster.callApi().onLeaderOverlord( - o -> o.taskStatuses(Set.of(UNKNOWN_TASK_ID)) + overlord -> overlord.taskStatuses(Set.of(UNKNOWN_TASK_ID)) ); Assertions.assertTrue(result.isEmpty()); } @@ -191,7 +187,7 @@ public void test_taskStatuses_byIds_returnsEmpty_forUnknownTaskIds() public void test_taskStatus_fails_forUnknownTaskId() { verifyApiFailsWith( - cluster.leaderOverlord().taskStatus(UNKNOWN_TASK_ID), + overlord -> overlord.taskStatus(UNKNOWN_TASK_ID), UNKNOWN_TASK_ERROR ); } @@ -200,7 +196,7 @@ public void test_taskStatus_fails_forUnknownTaskId() public void test_taskPayload_fails_forUnknownTaskId() { verifyApiFailsWith( - cluster.leaderOverlord().taskPayload(UNKNOWN_TASK_ID), + overlord -> overlord.taskPayload(UNKNOWN_TASK_ID), UNKNOWN_TASK_ERROR ); } @@ -256,7 +252,7 @@ public void test_postSupervisor_fails_ifRequiredExtensionIsNotLoaded() public void test_findLockedIntervals_fails_whenNoFilter() { verifyApiFailsWith( - cluster.leaderOverlord().findLockedIntervals(List.of()), + o -> o.findLockedIntervals(List.of()), "No filter provided" ); } @@ -354,38 +350,12 @@ public void test_markSegmentAsUnused() Assertions.assertNotNull(result); } - private static void verifyApiFailsWith(ListenableFuture future, String message) + private void verifyApiFailsWith(Function> overlordApi, String message) { - final CountDownLatch isFutureDone = new CountDownLatch(1); - final AtomicReference capturedError = new AtomicReference<>(); - Futures.addCallback( - future, - new FutureCallback<>() - { - @Override - public void onSuccess(T result) - { - isFutureDone.countDown(); - } - - @Override - public void onFailure(Throwable t) - { - capturedError.set(t); - isFutureDone.countDown(); - } - }, - MoreExecutors.directExecutor() + Exception exception = Assertions.assertThrows( + Exception.class, + () -> cluster.callApi().onLeaderOverlord(overlordApi) ); - - try { - isFutureDone.await(); - } - catch (InterruptedException e) { - throw new RuntimeException(e); - } - - Assertions.assertNotNull(capturedError.get()); - Assertions.assertTrue(capturedError.get().getMessage().contains(message)); + Assertions.assertTrue(exception.getMessage().contains(message)); } } diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/TaskBuilder.java b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/TaskBuilder.java index e616471034c0..f6e0e29f7d97 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/TaskBuilder.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/TaskBuilder.java @@ -41,7 +41,6 @@ import org.apache.druid.indexing.common.task.batch.parallel.ParallelIndexSupervisorTask; import org.apache.druid.indexing.common.task.batch.parallel.ParallelIndexTuningConfig; import org.apache.druid.indexing.input.DruidInputSource; -import org.apache.druid.java.util.common.ISE; import org.apache.druid.java.util.common.granularity.Granularity; import org.apache.druid.query.aggregation.AggregatorFactory; import org.apache.druid.segment.TestHelper; @@ -49,8 +48,6 @@ import org.joda.time.Interval; import java.io.File; -import java.net.URL; -import java.util.ArrayList; import java.util.HashMap; import java.util.List; import java.util.Map; @@ -156,26 +153,11 @@ public B druidInputSource(String dataSource, Interval interval) * } * */ - public B localInputSourceWithFiles(String... resources) + public B localInputSourceWithFiles(File... files) { - try { - final List files = new ArrayList<>(); - for (String file : resources) { - final URL resourceUrl = getClass().getClassLoader().getResource(file); - if (resourceUrl == null) { - throw new ISE("Could not find file[%s]", file); - } - - files.add(new File(resourceUrl.toURI())); - } - - return inputSource( - new LocalInputSource(null, null, files, null) - ); - } - catch (Exception e) { - throw new RuntimeException(e); - } + return inputSource( + new LocalInputSource(null, null, List.of(files), null) + ); } public B inputFormat(InputFormat inputFormat) diff --git a/integration-tests-ex/cases/cluster/Security/docker-compose.py b/integration-tests-ex/cases/cluster/Security/docker-compose.py deleted file mode 100644 index 9ecb40de19e3..000000000000 --- a/integration-tests-ex/cases/cluster/Security/docker-compose.py +++ /dev/null @@ -1,36 +0,0 @@ -# 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. - -from template import BaseTemplate, generate - -class Template(BaseTemplate): - - def define_indexer(self): - service = super().define_indexer() - self.add_property(service, 'druid.msq.intermediate.storage.enable', 'true') - self.add_property(service, 'druid.msq.intermediate.storage.type', 'local') - self.add_property(service, 'druid.msq.intermediate.storage.basePath', '/shared/durablestorage/') - self.add_property(service, 'druid.export.storage.baseDir', '/') - - # No kafka dependency in this cluster - def define_kafka(self): - pass - - def extend_druid_service(self, service): - self.add_env_file(service, '../Common/environment-configs/auth.env') - self.add_env(service, 'druid_test_loadList', 'druid-basic-security') - - -generate(__file__, Template()) diff --git a/integration-tests-ex/cases/pom.xml b/integration-tests-ex/cases/pom.xml index 36ed8b439086..9c493ec2b3e0 100644 --- a/integration-tests-ex/cases/pom.xml +++ b/integration-tests-ex/cases/pom.xml @@ -554,15 +554,6 @@ - - IT-Security - - false - - - Security - - IT-DruidExactCountBitmap diff --git a/integration-tests-ex/cases/src/test/java/org/apache/druid/testsEx/categories/Security.java b/integration-tests-ex/cases/src/test/java/org/apache/druid/testsEx/categories/Security.java deleted file mode 100644 index 07054e1f4204..000000000000 --- a/integration-tests-ex/cases/src/test/java/org/apache/druid/testsEx/categories/Security.java +++ /dev/null @@ -1,24 +0,0 @@ -/* - * 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.testsEx.categories; - -public class Security -{ -} 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 deleted file mode 100644 index 7ba18e5a98f3..000000000000 --- a/integration-tests-ex/cases/src/test/resources/indexer/export_task.json +++ /dev/null @@ -1,222 +0,0 @@ -{ - "type": "query_controller", - "id": "%%QUERY_ID%%", - "spec": { - "query": { - "queryType": "scan", - "dataSource": { - "type": "external", - "inputSource": { - "type": "local", - "files": [ - "/resources/data/batch_index/json/wikipedia_index_data1.json" - ] - }, - "inputFormat": { - "type": "json", - "keepNullColumns": false, - "assumeNewlineDelimited": false, - "useJsonNodeReader": false - }, - "signature": [ - { - "name": "timestamp", - "type": "STRING" - }, - { - "name": "isRobot", - "type": "STRING" - }, - { - "name": "diffUrl", - "type": "STRING" - }, - { - "name": "added", - "type": "LONG" - }, - { - "name": "countryIsoCode", - "type": "STRING" - }, - { - "name": "regionName", - "type": "STRING" - }, - { - "name": "channel", - "type": "STRING" - }, - { - "name": "flags", - "type": "STRING" - }, - { - "name": "delta", - "type": "LONG" - }, - { - "name": "isUnpatrolled", - "type": "STRING" - }, - { - "name": "isNew", - "type": "STRING" - }, - { - "name": "deltaBucket", - "type": "DOUBLE" - }, - { - "name": "isMinor", - "type": "STRING" - }, - { - "name": "isAnonymous", - "type": "STRING" - }, - { - "name": "deleted", - "type": "LONG" - }, - { - "name": "cityName", - "type": "STRING" - }, - { - "name": "metroCode", - "type": "LONG" - }, - { - "name": "namespace", - "type": "STRING" - }, - { - "name": "comment", - "type": "STRING" - }, - { - "name": "page", - "type": "STRING" - }, - { - "name": "commentLength", - "type": "LONG" - }, - { - "name": "countryName", - "type": "STRING" - }, - { - "name": "user", - "type": "STRING" - }, - { - "name": "regionIsoCode", - "type": "STRING" - } - ] - }, - "intervals": { - "type": "intervals", - "intervals": [ - "-146136543-09-08T08:23:32.096Z/146140482-04-24T15:36:27.903Z" - ] - }, - "resultFormat": "compactedList", - "columns": [ - "added", - "delta", - "page" - ], - "context": { - "__exportFileFormat": "CSV", - "__resultFormat": "array", - "__user": "allowAll", - "executionMode": "async", - "finalize": false, - "finalizeAggregations": false, - "groupByEnableMultiValueUnnesting": false, - "maxNumTasks": 4, - "maxParseExceptions": 0, - "queryId": "b1491ce2-7d2a-4a7a-baa6-25a1a77135e5", - "scanSignature": "[{\"name\":\"added\",\"type\":\"LONG\"},{\"name\":\"delta\",\"type\":\"LONG\"},{\"name\":\"page\",\"type\":\"STRING\"}]", - "sqlQueryId": "b1491ce2-7d2a-4a7a-baa6-25a1a77135e5", - "sqlStringifyArrays": false, - "waitUntilSegmentsLoad": true - }, - "columnTypes": [ - "LONG", - "LONG", - "STRING" - ], - "granularity": { - "type": "all" - } - }, - "columnMappings": [ - { - "queryColumn": "page", - "outputColumn": "page" - }, - { - "queryColumn": "added", - "outputColumn": "added" - }, - { - "queryColumn": "delta", - "outputColumn": "delta" - } - ], - "destination": { - "type": "export", - "exportStorageProvider": { - "type": "local", - "exportPath": "/shared/export/" - }, - "resultFormat": "csv" - }, - "assignmentStrategy": "max", - "tuningConfig": { - "maxNumWorkers": 3, - "maxRowsInMemory": 100000, - "rowsPerSegment": 3000000 - } - }, - "sqlQuery": " INSERT INTO extern(local(exportPath => '/shared/export/'))\n AS CSV\n SELECT page, added, delta\n FROM TABLE(\n EXTERN(\n '{\"type\":\"local\",\"files\":[\"/resources/data/batch_index/json/wikipedia_index_data1.json\"]}',\n '{\"type\":\"json\"}',\n '[{\"type\":\"string\",\"name\":\"timestamp\"},{\"type\":\"string\",\"name\":\"isRobot\"},{\"type\":\"string\",\"name\":\"diffUrl\"},{\"type\":\"long\",\"name\":\"added\"},{\"type\":\"string\",\"name\":\"countryIsoCode\"},{\"type\":\"string\",\"name\":\"regionName\"},{\"type\":\"string\",\"name\":\"channel\"},{\"type\":\"string\",\"name\":\"flags\"},{\"type\":\"long\",\"name\":\"delta\"},{\"type\":\"string\",\"name\":\"isUnpatrolled\"},{\"type\":\"string\",\"name\":\"isNew\"},{\"type\":\"double\",\"name\":\"deltaBucket\"},{\"type\":\"string\",\"name\":\"isMinor\"},{\"type\":\"string\",\"name\":\"isAnonymous\"},{\"type\":\"long\",\"name\":\"deleted\"},{\"type\":\"string\",\"name\":\"cityName\"},{\"type\":\"long\",\"name\":\"metroCode\"},{\"type\":\"string\",\"name\":\"namespace\"},{\"type\":\"string\",\"name\":\"comment\"},{\"type\":\"string\",\"name\":\"page\"},{\"type\":\"long\",\"name\":\"commentLength\"},{\"type\":\"string\",\"name\":\"countryName\"},{\"type\":\"string\",\"name\":\"user\"},{\"type\":\"string\",\"name\":\"regionIsoCode\"}]'\n )\n )\n", - "sqlQueryContext": { - "__exportFileFormat": "CSV", - "finalizeAggregations": false, - "sqlQueryId": "b1491ce2-7d2a-4a7a-baa6-25a1a77135e5", - "groupByEnableMultiValueUnnesting": false, - "maxNumTasks": 4, - "waitUntilSegmentsLoad": true, - "executionMode": "async", - "__resultFormat": "array", - "sqlStringifyArrays": false, - "queryId": "b1491ce2-7d2a-4a7a-baa6-25a1a77135e5" - }, - "sqlResultsContext": { - "timeZone": "UTC", - "stringifyArrays": false - }, - "sqlTypeNames": [ - "VARCHAR", - "BIGINT", - "BIGINT" - ], - "nativeTypeNames": [ - "STRING", - "LONG", - "LONG" - ], - "context": { - "forceTimeChunkLock": true - }, - "groupId": "%%QUERY_ID%%", - "dataSource": "__query_export", - "resource": { - "availabilityGroup": "%%QUERY_ID%%", - "requiredCapacity": 1 - } -} diff --git a/integration-tests/src/main/java/org/apache/druid/testing/clients/SecurityClient.java b/integration-tests/src/main/java/org/apache/druid/testing/clients/SecurityClient.java deleted file mode 100644 index 5d8075bd696a..000000000000 --- a/integration-tests/src/main/java/org/apache/druid/testing/clients/SecurityClient.java +++ /dev/null @@ -1,238 +0,0 @@ -/* - * 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.testing.clients; - -import com.fasterxml.jackson.databind.ObjectMapper; -import com.google.common.collect.ImmutableMap; -import com.google.inject.Inject; -import org.apache.druid.java.util.common.ISE; -import org.apache.druid.java.util.common.StringUtils; -import org.apache.druid.java.util.http.client.HttpClient; -import org.apache.druid.java.util.http.client.Request; -import org.apache.druid.java.util.http.client.response.StatusResponseHandler; -import org.apache.druid.java.util.http.client.response.StatusResponseHolder; -import org.apache.druid.server.security.ResourceAction; -import org.apache.druid.testing.IntegrationTestingConfig; -import org.jboss.netty.handler.codec.http.HttpMethod; -import org.jboss.netty.handler.codec.http.HttpResponseStatus; -import org.testng.Assert; - -import javax.ws.rs.core.MediaType; -import java.io.IOException; -import java.net.URL; -import java.util.List; - -public class SecurityClient -{ - private final ObjectMapper jsonMapper; - private final HttpClient httpClient; - private final String coordinator; - private final StatusResponseHandler responseHandler; - - @Inject - SecurityClient( - ObjectMapper jsonMapper, - @AdminClient HttpClient httpClient, - IntegrationTestingConfig config - ) - { - this.jsonMapper = jsonMapper; - this.httpClient = httpClient; - this.coordinator = config.getCoordinatorUrl(); - this.responseHandler = StatusResponseHandler.getInstance(); - } - - public void createAuthenticationUser(String username) throws IOException - { - final Request request = new Request( - HttpMethod.POST, - new URL( - StringUtils.format( - "%s/users/%s", - getAuthenticatorURL(), - StringUtils.urlEncode(username) - ) - ) - ); - Assert.assertEquals(HttpResponseStatus.OK, sendRequest(request).getStatus()); - } - - public void deleteAuthenticationUser(String username) throws IOException - { - final Request request = new Request( - HttpMethod.DELETE, - new URL( - StringUtils.format( - "%s/users/%s", - getAuthenticatorURL(), - StringUtils.urlEncode(username) - ) - ) - ); - Assert.assertEquals(HttpResponseStatus.OK, sendRequest(request).getStatus()); - } - - public void setUserPassword(String username, String password) throws IOException - { - final Request request = new Request( - HttpMethod.POST, - new URL( - StringUtils.format( - "%s/users/%s/credentials", - getAuthenticatorURL(), - StringUtils.urlEncode(username) - ) - ) - ); - - request.setContent(MediaType.APPLICATION_JSON, jsonMapper.writeValueAsBytes(ImmutableMap.of("password", password))); - Assert.assertEquals(HttpResponseStatus.OK, sendRequest(request).getStatus()); - } - - public void createAuthorizerUser(String username) throws IOException - { - final Request request = new Request( - HttpMethod.POST, - new URL( - StringUtils.format( - "%s/users/%s", - getAuthorizerURL(), - StringUtils.urlEncode(username) - ) - ) - ); - Assert.assertEquals(HttpResponseStatus.OK, sendRequest(request).getStatus()); - } - - public void deleteAuthorizerUser(String username) throws IOException - { - final Request request = new Request( - HttpMethod.DELETE, - new URL( - StringUtils.format( - "%s/users/%s", - getAuthorizerURL(), - StringUtils.urlEncode(username) - ) - ) - ); - Assert.assertEquals(HttpResponseStatus.OK, sendRequest(request).getStatus()); - } - - public void createAuthorizerRole(String role) throws IOException - { - final Request request = new Request( - HttpMethod.POST, - new URL( - StringUtils.format( - "%s/roles/%s", - getAuthorizerURL(), - StringUtils.urlEncode(role) - ) - ) - ); - Assert.assertEquals(HttpResponseStatus.OK, sendRequest(request).getStatus()); - } - - public void deleteAuthorizerRole(String role) throws IOException - { - final Request request = new Request( - HttpMethod.DELETE, - new URL( - StringUtils.format( - "%s/roles/%s", - getAuthorizerURL(), - StringUtils.urlEncode(role) - ) - ) - ); - Assert.assertEquals(HttpResponseStatus.OK, sendRequest(request).getStatus()); - } - - public void assignUserToRole(String user, String role) throws IOException - { - final Request request = new Request( - HttpMethod.POST, - new URL( - StringUtils.format( - "%s/users/%s/roles/%s", - getAuthorizerURL(), - StringUtils.urlEncode(user), - StringUtils.urlEncode(role) - ) - ) - ); - Assert.assertEquals(HttpResponseStatus.OK, sendRequest(request).getStatus()); - } - - public void setPermissionsToRole(String role, List permissions) throws IOException - { - final Request request = new Request( - HttpMethod.POST, - new URL( - StringUtils.format( - "%s/roles/%s/permissions/", - getAuthorizerURL(), - StringUtils.urlEncode(role) - ) - ) - ).setContent(MediaType.APPLICATION_JSON, jsonMapper.writeValueAsBytes(permissions)); - Assert.assertEquals(HttpResponseStatus.OK, sendRequest(request).getStatus()); - } - - private StatusResponseHolder sendRequest(Request request) - { - try { - final StatusResponseHolder response = httpClient.go( - request, - responseHandler - ).get(); - - if (!response.getStatus().equals(HttpResponseStatus.OK)) { - throw new ISE( - "Error while creating users status [%s] content [%s]", - response.getStatus(), - response.getContent() - ); - } - - return response; - } - catch (Exception e) { - throw new RuntimeException(e); - } - } - - private String getAuthenticatorURL() - { - return StringUtils.format( - "%s/druid-ext/basic-security/authentication/db/basic", - coordinator - ); - } - - private String getAuthorizerURL() - { - return StringUtils.format( - "%s/druid-ext/basic-security/authorization/db/basic", - coordinator - ); - } -} diff --git a/server/src/main/java/org/apache/druid/client/coordinator/CoordinatorServiceClient.java b/server/src/main/java/org/apache/druid/client/coordinator/CoordinatorServiceClient.java index a50d9b19862a..1e15ea445462 100644 --- a/server/src/main/java/org/apache/druid/client/coordinator/CoordinatorServiceClient.java +++ b/server/src/main/java/org/apache/druid/client/coordinator/CoordinatorServiceClient.java @@ -19,11 +19,7 @@ package org.apache.druid.client.coordinator; -import org.apache.druid.discovery.NodeRole; import org.apache.druid.rpc.ServiceClient; -import org.apache.druid.rpc.ServiceClientFactory; -import org.apache.druid.rpc.ServiceLocator; -import org.apache.druid.rpc.StandardRetryPolicy; /** * Wrapper over {@link ServiceClient} used to send requests to the Coordinator. @@ -36,16 +32,10 @@ public class CoordinatorServiceClient private final ServiceClient serviceClient; public CoordinatorServiceClient( - ServiceClientFactory clientFactory, - ServiceLocator coordinatorServiceLocator, - int maxRetryAttempts + ServiceClient serviceClient ) { - this.serviceClient = clientFactory.makeClient( - NodeRole.COORDINATOR.getJsonName(), - coordinatorServiceLocator, - StandardRetryPolicy.builder().maxAttempts(maxRetryAttempts).build() - ); + this.serviceClient = serviceClient; } public ServiceClient getServiceClient() diff --git a/server/src/main/java/org/apache/druid/rpc/guice/ServiceClientModule.java b/server/src/main/java/org/apache/druid/rpc/guice/ServiceClientModule.java index d99020af708b..1accdfb52ab6 100644 --- a/server/src/main/java/org/apache/druid/rpc/guice/ServiceClientModule.java +++ b/server/src/main/java/org/apache/druid/rpc/guice/ServiceClientModule.java @@ -40,6 +40,7 @@ import org.apache.druid.java.util.common.concurrent.ScheduledExecutors; import org.apache.druid.java.util.http.client.HttpClient; import org.apache.druid.rpc.DiscoveryServiceLocator; +import org.apache.druid.rpc.ServiceClient; import org.apache.druid.rpc.ServiceClientFactory; import org.apache.druid.rpc.ServiceClientFactoryImpl; import org.apache.druid.rpc.ServiceLocator; @@ -77,23 +78,29 @@ public ServiceLocator makeOverlordServiceLocator(final DruidNodeDiscoveryProvide } @Provides - @LazySingleton - public OverlordClient makeOverlordClient( - @Json final ObjectMapper jsonMapper, + @IndexingService + public ServiceClient makeServiceClientForOverlord( @EscalatedGlobal final ServiceClientFactory clientFactory, @IndexingService final ServiceLocator serviceLocator ) { - return new OverlordClientImpl( - clientFactory.makeClient( - NodeRole.OVERLORD.getJsonName(), - serviceLocator, - StandardRetryPolicy.builder().maxAttempts(CLIENT_MAX_ATTEMPTS).build() - ), - jsonMapper + return clientFactory.makeClient( + NodeRole.OVERLORD.getJsonName(), + serviceLocator, + StandardRetryPolicy.builder().maxAttempts(CLIENT_MAX_ATTEMPTS).build() ); } + @Provides + @LazySingleton + public OverlordClient makeOverlordClient( + @Json final ObjectMapper jsonMapper, + @IndexingService final ServiceClient serviceClient + ) + { + return new OverlordClientImpl(serviceClient, jsonMapper); + } + @Provides @ManageLifecycle @Coordinator @@ -103,23 +110,29 @@ public ServiceLocator makeCoordinatorServiceLocator(final DruidNodeDiscoveryProv } @Provides - @LazySingleton - public CoordinatorClient makeCoordinatorClient( - @Json final ObjectMapper jsonMapper, + @Coordinator + public ServiceClient makeServiceClientForCoordinator( @EscalatedGlobal final ServiceClientFactory clientFactory, @Coordinator final ServiceLocator serviceLocator ) { - return new CoordinatorClientImpl( - clientFactory.makeClient( - NodeRole.COORDINATOR.getJsonName(), - serviceLocator, - StandardRetryPolicy.builder().maxAttempts(CLIENT_MAX_ATTEMPTS).build() - ), - jsonMapper + return clientFactory.makeClient( + NodeRole.COORDINATOR.getJsonName(), + serviceLocator, + StandardRetryPolicy.builder().maxAttempts(CLIENT_MAX_ATTEMPTS).build() ); } + @Provides + @LazySingleton + public CoordinatorClient makeCoordinatorClient( + @Json final ObjectMapper jsonMapper, + @Coordinator final ServiceClient serviceClient + ) + { + return new CoordinatorClientImpl(serviceClient, jsonMapper); + } + /** * Creates a {@link CoordinatorServiceClient} used by extensions to send * requests to the Coordinator. For core Coordinator APIs, @@ -128,11 +141,10 @@ public CoordinatorClient makeCoordinatorClient( @Provides @LazySingleton public static CoordinatorServiceClient createCoordinatorServiceClient( - @EscalatedGlobal ServiceClientFactory clientFactory, - @Coordinator ServiceLocator serviceLocator + @Coordinator final ServiceClient serviceClient ) { - return new CoordinatorServiceClient(clientFactory, serviceLocator, CLIENT_MAX_ATTEMPTS); + return new CoordinatorServiceClient(serviceClient); } @Provides @@ -144,23 +156,29 @@ public ServiceLocator makeBrokerServiceLocator(final DruidNodeDiscoveryProvider } @Provides - @LazySingleton - public BrokerClient makeBrokerClient( - @Json final ObjectMapper jsonMapper, + @Broker + public ServiceClient makeServiceClientForBroker( @EscalatedGlobal final ServiceClientFactory clientFactory, @Broker final ServiceLocator serviceLocator ) { - return new BrokerClientImpl( - clientFactory.makeClient( - NodeRole.BROKER.getJsonName(), - serviceLocator, - StandardRetryPolicy.builder().maxAttempts(ServiceClientModule.CLIENT_MAX_ATTEMPTS).build() - ), - jsonMapper + return clientFactory.makeClient( + NodeRole.BROKER.getJsonName(), + serviceLocator, + StandardRetryPolicy.builder().maxAttempts(ServiceClientModule.CLIENT_MAX_ATTEMPTS).build() ); } + @Provides + @LazySingleton + public BrokerClient makeBrokerClient( + @Json final ObjectMapper jsonMapper, + @Broker final ServiceClient serviceClient + ) + { + return new BrokerClientImpl(serviceClient, jsonMapper); + } + public static ServiceClientFactory makeServiceClientFactory(@EscalatedGlobal final HttpClient httpClient) { final ScheduledExecutorService connectExec = diff --git a/services/src/test/java/org/apache/druid/testing/embedded/EmbeddedClusterApis.java b/services/src/test/java/org/apache/druid/testing/embedded/EmbeddedClusterApis.java index 954e0f117c43..0cd079a19a72 100644 --- a/services/src/test/java/org/apache/druid/testing/embedded/EmbeddedClusterApis.java +++ b/services/src/test/java/org/apache/druid/testing/embedded/EmbeddedClusterApis.java @@ -22,9 +22,9 @@ import com.fasterxml.jackson.core.type.TypeReference; import com.google.common.collect.ImmutableList; import com.google.common.util.concurrent.ListenableFuture; +import org.apache.druid.client.broker.BrokerClient; import org.apache.druid.client.coordinator.CoordinatorClient; import org.apache.druid.client.indexing.TaskStatusResponse; -import org.apache.druid.common.guava.FutureUtils; import org.apache.druid.common.utils.IdUtils; import org.apache.druid.indexer.TaskState; import org.apache.druid.indexer.TaskStatus; @@ -66,28 +66,57 @@ * @see #onLeaderOverlord(Function) * @see #runSql(String, Object...) */ -public class EmbeddedClusterApis +public class EmbeddedClusterApis implements EmbeddedResource { private final EmbeddedDruidCluster cluster; + private EmbeddedServiceClient clients; EmbeddedClusterApis(EmbeddedDruidCluster cluster) { this.cluster = cluster; } + @Override + public void start() throws Exception + { + this.clients = EmbeddedServiceClient.create(cluster, null); + } + + @Override + public void stop() throws Exception + { + if (clients != null) { + clients.stop(); + clients = null; + } + } + + public EmbeddedServiceClient serviceClients() + { + return Objects.requireNonNull( + clients, + "Service clients are not initialized. Ensure that the cluster has started properly." + ); + } + public T onLeaderCoordinator(Function> coordinatorApi) { - return getResult(coordinatorApi.apply(cluster.leaderCoordinator())); + return clients.onLeaderCoordinator(coordinatorApi); } public T onLeaderCoordinatorSync(Function coordinatorApi) { - return coordinatorApi.apply(cluster.leaderCoordinator()); + return clients.onLeaderCoordinatorSync(coordinatorApi); } public T onLeaderOverlord(Function> overlordApi) { - return getResult(overlordApi.apply(cluster.leaderOverlord())); + return clients.onLeaderOverlord(overlordApi); + } + + public T onAnyBroker(Function> brokerApi) + { + return clients.onAnyBroker(brokerApi); } /** @@ -99,8 +128,8 @@ public T onLeaderOverlord(Function> over public String runSql(String sql, Object... args) { try { - return getResult( - cluster.anyBroker().submitSqlQuery( + return onAnyBroker( + b -> b.submitSqlQuery( new ClientSqlQuery( StringUtils.format(sql, args), ResultFormat.CSV.name(), @@ -162,9 +191,11 @@ public void submitTask(Task task) */ public void waitForTaskToSucceed(String taskId, EmbeddedOverlord overlord) { + TaskStatus taskStatus = waitForTaskToFinish(taskId, overlord); Assertions.assertEquals( TaskState.SUCCESS, - waitForTaskToFinish(taskId, overlord).getStatusCode() + taskStatus.getStatusCode(), + StringUtils.format("Task[%s] failed with error[%s]", taskId, taskStatus.getErrorMsg()) ); } @@ -380,11 +411,6 @@ public static List createAlignedIntervals( return alignedIntervals; } - private static T getResult(ListenableFuture future) - { - return FutureUtils.getUnchecked(future, true); - } - @FunctionalInterface public interface TaskBuilder { diff --git a/services/src/test/java/org/apache/druid/testing/embedded/EmbeddedDruidCluster.java b/services/src/test/java/org/apache/druid/testing/embedded/EmbeddedDruidCluster.java index 3fb2cba18bc5..db025f20da72 100644 --- a/services/src/test/java/org/apache/druid/testing/embedded/EmbeddedDruidCluster.java +++ b/services/src/test/java/org/apache/druid/testing/embedded/EmbeddedDruidCluster.java @@ -21,13 +21,10 @@ import com.google.common.base.Preconditions; import com.google.common.collect.Lists; -import org.apache.druid.client.broker.BrokerClient; -import org.apache.druid.client.coordinator.CoordinatorClient; import org.apache.druid.initialization.DruidModule; 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.rpc.indexing.OverlordClient; import org.apache.druid.server.metrics.LatchableEmitter; import org.apache.druid.testing.embedded.derby.InMemoryDerbyModule; import org.apache.druid.testing.embedded.derby.InMemoryDerbyResource; @@ -71,11 +68,11 @@ * cluster.stop(); * */ -public class EmbeddedDruidCluster implements ClusterReferencesProvider, EmbeddedResource +public class EmbeddedDruidCluster implements EmbeddedResource { private static final Logger log = new Logger(EmbeddedDruidCluster.class); - private final EmbeddedClusterApis clusterApis; + private final EmbeddedClusterApis clusterApis = new EmbeddedClusterApis(this); private final TestFolder testFolder = new TestFolder(); private final List> servers = new ArrayList<>(); @@ -89,7 +86,6 @@ public class EmbeddedDruidCluster implements ClusterReferencesProvider, Embedded private EmbeddedDruidCluster() { resources.add(testFolder); - clusterApis = new EmbeddedClusterApis(this); addExtension(RuntimeInfoModule.class); } @@ -138,6 +134,18 @@ private void addEmbeddedZookeeper() resources.add(zookeeper); } + /** + * Adds the {@link #clusterApis} as the last entry in the {@link #resources} list. + * {@link EmbeddedClusterApis} must be started after all servers have started + * so that the injected mappers and clients can be used. + */ + private void addEmbeddedClusterApis() + { + if (!startedFirstDruidServer) { + resources.add(clusterApis); + } + } + /** * Configures this cluster to use a {@link LatchableEmitter}. This method is a * shorthand for the following: @@ -249,6 +257,7 @@ public EmbeddedZookeeper getZookeeper() public void start() throws Exception { Preconditions.checkArgument(!servers.isEmpty(), "Cluster must have at least one embedded Druid server"); + addEmbeddedClusterApis(); // Start the resources in order for (EmbeddedResource resource : resources) { @@ -314,25 +323,12 @@ public String runSql(String sql, Object... args) return clusterApis.runSql(sql, args); } - @Override - public CoordinatorClient leaderCoordinator() - { - return findServerOfType(EmbeddedCoordinator.class).bindings().leaderCoordinator(); - } - - @Override - public OverlordClient leaderOverlord() - { - return findServerOfType(EmbeddedOverlord.class).bindings().leaderOverlord(); - } - - @Override - public BrokerClient anyBroker() + EmbeddedDruidServer anyServer() { - return findServerOfType(EmbeddedBroker.class).bindings().anyBroker(); + return servers.get(0); } - private > EmbeddedDruidServer findServerOfType( + > S findServerOfType( Class serverType ) { diff --git a/services/src/test/java/org/apache/druid/testing/embedded/EmbeddedServerLifecycle.java b/services/src/test/java/org/apache/druid/testing/embedded/EmbeddedServerLifecycle.java index 8cc4948f10df..3795986676cc 100644 --- a/services/src/test/java/org/apache/druid/testing/embedded/EmbeddedServerLifecycle.java +++ b/services/src/test/java/org/apache/druid/testing/embedded/EmbeddedServerLifecycle.java @@ -76,7 +76,7 @@ public void start() throws Exception executorService.submit(() -> runServer(serverRunnable)); // Wait for lifecycle to be created and started - if (lifecycleCreated.await(10, TimeUnit.SECONDS)) { + if (lifecycleCreated.await(100, TimeUnit.SECONDS)) { awaitLifecycleStart(); } else { throw new ISE("Timed out waiting for lifecycle of server[%s] to be created", server.getName()); diff --git a/services/src/test/java/org/apache/druid/testing/embedded/EmbeddedServiceClient.java b/services/src/test/java/org/apache/druid/testing/embedded/EmbeddedServiceClient.java new file mode 100644 index 000000000000..a9b9aafca95a --- /dev/null +++ b/services/src/test/java/org/apache/druid/testing/embedded/EmbeddedServiceClient.java @@ -0,0 +1,218 @@ +/* + * 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.testing.embedded; + +import com.fasterxml.jackson.core.type.TypeReference; +import com.fasterxml.jackson.databind.ObjectMapper; +import com.google.common.base.Throwables; +import com.google.common.util.concurrent.ListenableFuture; +import org.apache.druid.client.broker.Broker; +import org.apache.druid.client.broker.BrokerClient; +import org.apache.druid.client.coordinator.Coordinator; +import org.apache.druid.client.coordinator.CoordinatorClient; +import org.apache.druid.client.indexing.IndexingService; +import org.apache.druid.common.guava.FutureUtils; +import org.apache.druid.java.util.common.ISE; +import org.apache.druid.java.util.common.concurrent.ScheduledExecutors; +import org.apache.druid.java.util.http.client.HttpClient; +import org.apache.druid.java.util.http.client.response.StatusResponseHandler; +import org.apache.druid.java.util.http.client.response.StatusResponseHolder; +import org.apache.druid.rpc.RequestBuilder; +import org.apache.druid.rpc.ServiceClient; +import org.apache.druid.rpc.ServiceClientFactory; +import org.apache.druid.rpc.ServiceClientFactoryImpl; +import org.apache.druid.rpc.ServiceLocator; +import org.apache.druid.rpc.guice.ServiceClientModule; +import org.apache.druid.rpc.indexing.OverlordClient; +import org.apache.druid.server.security.Escalator; +import org.jboss.netty.handler.codec.http.HttpResponseStatus; + +import javax.annotation.Nullable; +import java.util.concurrent.ScheduledExecutorService; +import java.util.concurrent.TimeUnit; +import java.util.function.Function; + +/** + * Contains {@link ServiceClient} objects to connect to various services in an + * embedded test cluster. + */ +public class EmbeddedServiceClient +{ + private final EmbeddedDruidCluster cluster; + private final ServiceClientModule module; + + private final ServiceClient coordinatorServiceClient; + private final ServiceClient overlordServiceClient; + private final ServiceClient brokerServiceClient; + + private final ScheduledExecutorService clientConnectExec; + private final StatusResponseHandler responseHandler; + + private EmbeddedServiceClient(EmbeddedDruidCluster cluster, Escalator escalator) + { + // Use a ServiceClientModule to create the various clients + this.module = new ServiceClientModule(); + this.cluster = cluster; + this.clientConnectExec = ScheduledExecutors.fixed(4, "ServiceClientFactory-%d"); + this.responseHandler = StatusResponseHandler.getInstance(); + + final EmbeddedDruidServer anyServer = cluster.anyServer(); + + final HttpClient escalatedClient = + escalator == null + ? anyServer.bindings().escalatedHttpClient() + : escalator.createEscalatedClient(anyServer.bindings().globalHttpClient()); + + // Create service clients + final ServiceClientFactory factory = new ServiceClientFactoryImpl(escalatedClient, clientConnectExec); + this.overlordServiceClient = module.makeServiceClientForOverlord( + factory, + anyServer.bindings().getInstance(ServiceLocator.class, IndexingService.class) + ); + + this.brokerServiceClient = module.makeServiceClientForBroker( + factory, + anyServer.bindings().getInstance(ServiceLocator.class, Broker.class) + ); + this.coordinatorServiceClient = module.makeServiceClientForCoordinator( + factory, + anyServer.bindings().getInstance(ServiceLocator.class, Coordinator.class) + ); + } + + /** + * Creates a client that uses the {@link Escalator} bound to the embedded servers. + */ + public static EmbeddedServiceClient create(EmbeddedDruidCluster cluster) + { + return new EmbeddedServiceClient(cluster, null); + } + + /** + * Creates a client using the specified {@link Escalator}. All requests made by this + * client will use the given escalator. + */ + public static EmbeddedServiceClient create(EmbeddedDruidCluster cluster, Escalator escalator) + { + return new EmbeddedServiceClient(cluster, escalator); + } + + public void stop() throws InterruptedException + { + clientConnectExec.shutdownNow(); + clientConnectExec.awaitTermination(1, TimeUnit.MINUTES); + } + + @Nullable + public T onLeaderCoordinator( + Function request, + TypeReference resultType + ) + { + return makeRequest(request, resultType, coordinatorServiceClient, getMapper(EmbeddedCoordinator.class)); + } + + public T onLeaderCoordinator(Function> coordinatorApi) + { + return getResult(coordinatorApi.apply(createCoordinatorClient())); + } + + public T onLeaderCoordinatorSync(Function coordinatorApi) + { + return coordinatorApi.apply(createCoordinatorClient()); + } + + public T onLeaderOverlord(Function> overlordApi) + { + return getResult(overlordApi.apply(createOverlordClient())); + } + + @Nullable + public T onLeaderOverlord( + Function request, + TypeReference resultType + ) + { + return makeRequest(request, resultType, overlordServiceClient, getMapper(EmbeddedOverlord.class)); + } + + public T onAnyBroker(Function> brokerApi) + { + return getResult(brokerApi.apply(createBrokerClient())); + } + + @Nullable + private T makeRequest( + Function request, + TypeReference resultType, + ServiceClient serviceClient, + ObjectMapper mapper + ) + { + final RequestBuilder requestBuilder = request.apply(mapper); + + try { + StatusResponseHolder response = serviceClient.request(requestBuilder, responseHandler); + if (!response.getStatus().equals(HttpResponseStatus.OK)) { + throw new ISE( + "Request[%s] failed with status[%s] content[%s].", + requestBuilder.toString(), + response.getStatus(), + response.getContent() + ); + } + + if (resultType == null) { + return null; + } else { + return mapper.readValue(response.getContent(), resultType); + } + } + catch (Exception e) { + Throwables.throwIfUnchecked(e); + throw new RuntimeException(e); + } + } + + private CoordinatorClient createCoordinatorClient() + { + return module.makeCoordinatorClient(getMapper(EmbeddedCoordinator.class), coordinatorServiceClient); + } + + private OverlordClient createOverlordClient() + { + return module.makeOverlordClient(getMapper(EmbeddedOverlord.class), overlordServiceClient); + } + + private BrokerClient createBrokerClient() + { + return module.makeBrokerClient(getMapper(EmbeddedBroker.class), brokerServiceClient); + } + + private > ObjectMapper getMapper(Class serverType) + { + return cluster.findServerOfType(serverType).bindings().jsonMapper(); + } + + private static T getResult(ListenableFuture future) + { + return FutureUtils.getUnchecked(future, true); + } +} diff --git a/services/src/test/java/org/apache/druid/testing/embedded/ServerReferenceHolder.java b/services/src/test/java/org/apache/druid/testing/embedded/ServerReferenceHolder.java index 2e878621afe8..95966e8e3903 100644 --- a/services/src/test/java/org/apache/druid/testing/embedded/ServerReferenceHolder.java +++ b/services/src/test/java/org/apache/druid/testing/embedded/ServerReferenceHolder.java @@ -22,6 +22,7 @@ import com.fasterxml.jackson.databind.ObjectMapper; import com.google.inject.Inject; import com.google.inject.Injector; +import com.google.inject.Key; import org.apache.druid.client.broker.BrokerClient; import org.apache.druid.client.coordinator.Coordinator; import org.apache.druid.client.coordinator.CoordinatorClient; @@ -29,6 +30,7 @@ import org.apache.druid.discovery.DruidLeaderSelector; import org.apache.druid.discovery.DruidNodeDiscoveryProvider; import org.apache.druid.guice.annotations.EscalatedGlobal; +import org.apache.druid.guice.annotations.Global; import org.apache.druid.guice.annotations.Json; import org.apache.druid.guice.annotations.Self; import org.apache.druid.indexing.overlord.IndexerMetadataStorageCoordinator; @@ -38,6 +40,7 @@ import org.apache.druid.server.DruidNode; import org.apache.druid.server.metrics.LatchableEmitter; +import java.lang.annotation.Annotation; import java.util.Objects; /** @@ -78,7 +81,11 @@ public final class ServerReferenceHolder implements ServerReferencesProvider @Inject @EscalatedGlobal - private HttpClient httpClient; + private HttpClient escalatedHttpClient; + + @Inject + @Global + private HttpClient globalHttpClient; @Self @Inject @@ -154,7 +161,13 @@ public DruidNodeDiscoveryProvider nodeDiscovery() @Override public HttpClient escalatedHttpClient() { - return httpClient; + return escalatedHttpClient; + } + + @Override + public HttpClient globalHttpClient() + { + return globalHttpClient; } @Override @@ -168,4 +181,10 @@ public T getInstance(Class clazz) { return injector.getInstance(clazz); } + + @Override + public T getInstance(Class type, Class annotationType) + { + return injector.getInstance(Key.get(type, annotationType)); + } } diff --git a/services/src/test/java/org/apache/druid/testing/embedded/ServerReferencesProvider.java b/services/src/test/java/org/apache/druid/testing/embedded/ServerReferencesProvider.java index ca24218e0fb5..a1aad14a08f6 100644 --- a/services/src/test/java/org/apache/druid/testing/embedded/ServerReferencesProvider.java +++ b/services/src/test/java/org/apache/druid/testing/embedded/ServerReferencesProvider.java @@ -32,6 +32,8 @@ import org.apache.druid.server.DruidNode; import org.apache.druid.server.metrics.LatchableEmitter; +import java.lang.annotation.Annotation; + /** * Provides a handle to the various objects used by an {@link EmbeddedDruidServer} * during an embedded cluster test. The returned references should be used for @@ -95,6 +97,11 @@ public interface ServerReferencesProvider */ HttpClient escalatedHttpClient(); + /** + * Non-escalated {@link HttpClient} used by this server to communicate with other Druid servers. + */ + HttpClient globalHttpClient(); + /** * {@link ObjectMapper} annotated with {@link Json}. */ @@ -105,4 +112,6 @@ public interface ServerReferencesProvider * The returned object must be used for read-only purposes. */ T getInstance(Class clazz); + + T getInstance(Class clazz, Class annotation); } diff --git a/services/src/test/java/org/apache/druid/testing/embedded/indexing/Resources.java b/services/src/test/java/org/apache/druid/testing/embedded/indexing/Resources.java index 7af3d3a7c7a2..9f6ae4713765 100644 --- a/services/src/test/java/org/apache/druid/testing/embedded/indexing/Resources.java +++ b/services/src/test/java/org/apache/druid/testing/embedded/indexing/Resources.java @@ -19,11 +19,36 @@ package org.apache.druid.testing.embedded.indexing; +import com.google.common.base.Throwables; +import org.apache.druid.java.util.common.ISE; + +import java.io.File; +import java.net.URL; + /** * Constants and utility methods used in embedded cluster tests. */ public class Resources { + /** + * Returns the {@link File} for the given local resource. + */ + public static File getFileForResource(String resourceName) + { + final URL resourceUrl = DataFile.class.getClassLoader().getResource(resourceName); + if (resourceUrl == null) { + throw new ISE("Could not find resource file[%s]", resourceName); + } + + try { + return new File(resourceUrl.toURI()); + } + catch (Exception e) { + Throwables.throwIfUnchecked(e); + throw new RuntimeException(e); + } + } + public static class InlineData { /** @@ -67,9 +92,20 @@ public static class InlineData public static class DataFile { - public static final String TINY_WIKI_1_JSON = "data/json/tiny_wiki_1.json"; - public static final String TINY_WIKI_2_JSON = "data/json/tiny_wiki_2.json"; - public static final String TINY_WIKI_3_JSON = "data/json/tiny_wiki_3.json"; + public static File tinyWiki1Json() + { + return getFileForResource("data/json/tiny_wiki_1.json"); + } + + public static File tinyWiki2Json() + { + return getFileForResource("data/json/tiny_wiki_2.json"); + } + + public static File tinyWiki3Json() + { + return getFileForResource("data/json/tiny_wiki_3.json"); + } } /** From 7ecb7b8b86ef6f3b3089e4e985bb4ea4f50ffa5e Mon Sep 17 00:00:00 2001 From: Kashif Faraz Date: Mon, 4 Aug 2025 13:21:21 +0530 Subject: [PATCH 02/10] Minor cleanup --- .../embedded/auth/BasicAuthMsqTest.java | 2 +- .../compact/CompactionResourceTestClient.java | 2 +- .../testing/embedded/EmbeddedClusterApis.java | 25 +++++++++++-------- .../embedded/EmbeddedDruidCluster.java | 19 +++++--------- .../embedded/EmbeddedServerLifecycle.java | 2 +- .../embedded/ServerReferencesProvider.java | 4 +++ 6 files changed, 27 insertions(+), 27 deletions(-) diff --git a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/auth/BasicAuthMsqTest.java b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/auth/BasicAuthMsqTest.java index 89a9be844468..0d5d061dd35b 100644 --- a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/auth/BasicAuthMsqTest.java +++ b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/auth/BasicAuthMsqTest.java @@ -146,7 +146,7 @@ public void setupClient() ); // Use the default set of clients for calling security APIs - securityClient = new SecurityClient(cluster.callApi().serviceClients()); + securityClient = new SecurityClient(cluster.callApi().serviceClient()); } @BeforeEach diff --git a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/compact/CompactionResourceTestClient.java b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/compact/CompactionResourceTestClient.java index 359966a2f28f..1f32b0ca62cc 100644 --- a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/compact/CompactionResourceTestClient.java +++ b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/compact/CompactionResourceTestClient.java @@ -51,7 +51,7 @@ public class CompactionResourceTestClient CompactionResourceTestClient(EmbeddedDruidCluster cluster) { - this.clients = cluster.callApi().serviceClients(); + this.clients = cluster.callApi().serviceClient(); } private String getCoordinatorURL() diff --git a/services/src/test/java/org/apache/druid/testing/embedded/EmbeddedClusterApis.java b/services/src/test/java/org/apache/druid/testing/embedded/EmbeddedClusterApis.java index 0cd079a19a72..878cc055ffdb 100644 --- a/services/src/test/java/org/apache/druid/testing/embedded/EmbeddedClusterApis.java +++ b/services/src/test/java/org/apache/druid/testing/embedded/EmbeddedClusterApis.java @@ -69,7 +69,7 @@ public class EmbeddedClusterApis implements EmbeddedResource { private final EmbeddedDruidCluster cluster; - private EmbeddedServiceClient clients; + private EmbeddedServiceClient client; EmbeddedClusterApis(EmbeddedDruidCluster cluster) { @@ -79,44 +79,47 @@ public class EmbeddedClusterApis implements EmbeddedResource @Override public void start() throws Exception { - this.clients = EmbeddedServiceClient.create(cluster, null); + this.client = EmbeddedServiceClient.create(cluster, null); } @Override public void stop() throws Exception { - if (clients != null) { - clients.stop(); - clients = null; + if (client != null) { + client.stop(); + client = null; } } - public EmbeddedServiceClient serviceClients() + /** + * Client used for all the API calls made by this {@link EmbeddedClusterApis}. + */ + public EmbeddedServiceClient serviceClient() { return Objects.requireNonNull( - clients, + client, "Service clients are not initialized. Ensure that the cluster has started properly." ); } public T onLeaderCoordinator(Function> coordinatorApi) { - return clients.onLeaderCoordinator(coordinatorApi); + return client.onLeaderCoordinator(coordinatorApi); } public T onLeaderCoordinatorSync(Function coordinatorApi) { - return clients.onLeaderCoordinatorSync(coordinatorApi); + return client.onLeaderCoordinatorSync(coordinatorApi); } public T onLeaderOverlord(Function> overlordApi) { - return clients.onLeaderOverlord(overlordApi); + return client.onLeaderOverlord(overlordApi); } public T onAnyBroker(Function> brokerApi) { - return clients.onAnyBroker(brokerApi); + return client.onAnyBroker(brokerApi); } /** diff --git a/services/src/test/java/org/apache/druid/testing/embedded/EmbeddedDruidCluster.java b/services/src/test/java/org/apache/druid/testing/embedded/EmbeddedDruidCluster.java index db025f20da72..e2edad2749cd 100644 --- a/services/src/test/java/org/apache/druid/testing/embedded/EmbeddedDruidCluster.java +++ b/services/src/test/java/org/apache/druid/testing/embedded/EmbeddedDruidCluster.java @@ -134,18 +134,6 @@ private void addEmbeddedZookeeper() resources.add(zookeeper); } - /** - * Adds the {@link #clusterApis} as the last entry in the {@link #resources} list. - * {@link EmbeddedClusterApis} must be started after all servers have started - * so that the injected mappers and clients can be used. - */ - private void addEmbeddedClusterApis() - { - if (!startedFirstDruidServer) { - resources.add(clusterApis); - } - } - /** * Configures this cluster to use a {@link LatchableEmitter}. This method is a * shorthand for the following: @@ -257,7 +245,12 @@ public EmbeddedZookeeper getZookeeper() public void start() throws Exception { Preconditions.checkArgument(!servers.isEmpty(), "Cluster must have at least one embedded Druid server"); - addEmbeddedClusterApis(); + + // Add clusterApis as the last entry in the resources list, so that the + // EmbeddedServiceClient is initialized after mappers have been injected into the servers + if (!startedFirstDruidServer) { + resources.add(clusterApis); + } // Start the resources in order for (EmbeddedResource resource : resources) { diff --git a/services/src/test/java/org/apache/druid/testing/embedded/EmbeddedServerLifecycle.java b/services/src/test/java/org/apache/druid/testing/embedded/EmbeddedServerLifecycle.java index 3795986676cc..8cc4948f10df 100644 --- a/services/src/test/java/org/apache/druid/testing/embedded/EmbeddedServerLifecycle.java +++ b/services/src/test/java/org/apache/druid/testing/embedded/EmbeddedServerLifecycle.java @@ -76,7 +76,7 @@ public void start() throws Exception executorService.submit(() -> runServer(serverRunnable)); // Wait for lifecycle to be created and started - if (lifecycleCreated.await(100, TimeUnit.SECONDS)) { + if (lifecycleCreated.await(10, TimeUnit.SECONDS)) { awaitLifecycleStart(); } else { throw new ISE("Timed out waiting for lifecycle of server[%s] to be created", server.getName()); diff --git a/services/src/test/java/org/apache/druid/testing/embedded/ServerReferencesProvider.java b/services/src/test/java/org/apache/druid/testing/embedded/ServerReferencesProvider.java index a1aad14a08f6..c9d746d565bb 100644 --- a/services/src/test/java/org/apache/druid/testing/embedded/ServerReferencesProvider.java +++ b/services/src/test/java/org/apache/druid/testing/embedded/ServerReferencesProvider.java @@ -113,5 +113,9 @@ public interface ServerReferencesProvider */ T getInstance(Class clazz); + /** + * Gets the injected instance of the object of the specified type. + * The returned object must be used for read-only purposes. + */ T getInstance(Class clazz, Class annotation); } From 97e4842440fa145816c5f5499377a2bf252ff5af Mon Sep 17 00:00:00 2001 From: Kashif Faraz Date: Mon, 4 Aug 2025 13:46:54 +0530 Subject: [PATCH 03/10] Remove CoordinatorServiceClient --- .../embedded/auth/BasicAuthMsqTest.java | 22 +++------ .../testing/embedded/auth/SecurityClient.java | 36 +++++++-------- ...PollingBasicAuthenticatorCacheManager.java | 9 ++-- ...torPollingBasicAuthorizerCacheManager.java | 11 ++--- ...ingBasicAuthenticatorCacheManagerTest.java | 9 ++-- ...ollingBasicAuthorizerCacheManagerTest.java | 10 ++--- .../coordinator/CoordinatorServiceClient.java | 45 ------------------- .../druid/rpc/guice/ServiceClientModule.java | 15 ------- .../rpc/guice/ServiceClientModuleTest.java | 20 ++++++++- .../embedded/EmbeddedServiceClient.java | 7 +++ 10 files changed, 64 insertions(+), 120 deletions(-) delete mode 100644 server/src/main/java/org/apache/druid/client/coordinator/CoordinatorServiceClient.java diff --git a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/auth/BasicAuthMsqTest.java b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/auth/BasicAuthMsqTest.java index 0d5d061dd35b..05e1ffe4516f 100644 --- a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/auth/BasicAuthMsqTest.java +++ b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/auth/BasicAuthMsqTest.java @@ -58,10 +58,8 @@ import org.apache.druid.testing.embedded.EmbeddedBroker; import org.apache.druid.testing.embedded.EmbeddedCoordinator; import org.apache.druid.testing.embedded.EmbeddedDruidCluster; -import org.apache.druid.testing.embedded.EmbeddedHistorical; import org.apache.druid.testing.embedded.EmbeddedIndexer; import org.apache.druid.testing.embedded.EmbeddedOverlord; -import org.apache.druid.testing.embedded.EmbeddedRouter; import org.apache.druid.testing.embedded.EmbeddedServiceClient; import org.apache.druid.testing.embedded.indexing.Resources; import org.apache.druid.testing.embedded.junit5.EmbeddedClusterTestBase; @@ -79,8 +77,6 @@ public class BasicAuthMsqTest extends EmbeddedClusterTestBase { - private SecurityClient securityClient; - public static final String USER_1 = "user1"; public static final String ROLE_1 = "role1"; public static final String USER_1_PASSWORD = "password1"; @@ -89,15 +85,13 @@ public class BasicAuthMsqTest extends EmbeddedClusterTestBase // underlying test cluster enough time to sync permissions and be ready when test execution starts. private static final int SYNC_SLEEP = 500; - protected final EmbeddedBroker broker = new EmbeddedBroker(); - protected final EmbeddedIndexer indexer = new EmbeddedIndexer() + private SecurityClient securityClient; + private EmbeddedServiceClient userClient; + + private final EmbeddedOverlord overlord = new EmbeddedOverlord(); + private final EmbeddedIndexer indexer = new EmbeddedIndexer() .setServerMemory(400_000_000) .addProperty("druid.worker.capacity", "2"); - protected final EmbeddedOverlord overlord = new EmbeddedOverlord(); - protected final EmbeddedHistorical historical = new EmbeddedHistorical(); - protected final EmbeddedCoordinator coordinator = new EmbeddedCoordinator(); - - private EmbeddedServiceClient userClient; private final MsqExportDirectory exportDirectory = new MsqExportDirectory(); @Override @@ -107,12 +101,10 @@ protected EmbeddedDruidCluster createCluster() .withEmbeddedDerbyAndZookeeper() .useLatchableEmitter() .addResource(exportDirectory) - .addServer(coordinator) + .addServer(new EmbeddedCoordinator()) .addServer(overlord) .addServer(indexer) - .addServer(historical) - .addServer(broker) - .addServer(new EmbeddedRouter()) + .addServer(new EmbeddedBroker()) .addExtensions( BasicSecurityDruidModule.class, MSQSqlModule.class, diff --git a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/auth/SecurityClient.java b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/auth/SecurityClient.java index 1c9da7210c84..104194f18abc 100644 --- a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/auth/SecurityClient.java +++ b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/auth/SecurityClient.java @@ -30,15 +30,19 @@ import java.util.Map; import java.util.function.Function; +/** + * Client to call various basic auth APIs on the Coordinator. + */ public class SecurityClient { - private final String coordinator; + private static final String AUTHENTICATOR_URL = "/druid-ext/basic-security/authentication/db/basic"; + private static final String AUTHORIZER_URL = "/druid-ext/basic-security/authorization/db/basic"; + private final EmbeddedServiceClient clients; SecurityClient(EmbeddedServiceClient clients) { this.clients = clients; - this.coordinator = "/druid/coordinator/v1"; } public void createAuthenticationUser(String username) @@ -47,7 +51,7 @@ public void createAuthenticationUser(String username) HttpMethod.POST, StringUtils.format( "%s/users/%s", - getAuthenticatorURL(), + AUTHENTICATOR_URL, StringUtils.urlEncode(username) ) ); @@ -60,7 +64,7 @@ public void deleteAuthenticationUser(String username) HttpMethod.DELETE, StringUtils.format( "%s/users/%s", - getAuthenticatorURL(), + AUTHENTICATOR_URL, StringUtils.urlEncode(username) ) ); @@ -73,7 +77,7 @@ public void setUserPassword(String username, String password) HttpMethod.POST, StringUtils.format( "%s/users/%s/credentials", - getAuthenticatorURL(), + AUTHENTICATOR_URL, StringUtils.urlEncode(username) ) ); @@ -87,7 +91,7 @@ public void createAuthorizerUser(String username) HttpMethod.POST, StringUtils.format( "%s/users/%s", - getAuthorizerURL(), + AUTHORIZER_URL, StringUtils.urlEncode(username) ) ); @@ -100,7 +104,7 @@ public void deleteAuthorizerUser(String username) HttpMethod.DELETE, StringUtils.format( "%s/users/%s", - getAuthorizerURL(), + AUTHORIZER_URL, StringUtils.urlEncode(username) ) ); @@ -113,7 +117,7 @@ public void createAuthorizerRole(String role) HttpMethod.POST, StringUtils.format( "%s/roles/%s", - getAuthorizerURL(), + AUTHORIZER_URL, StringUtils.urlEncode(role) ) ); @@ -126,7 +130,7 @@ public void deleteAuthorizerRole(String role) HttpMethod.DELETE, StringUtils.format( "%s/roles/%s", - getAuthorizerURL(), + AUTHORIZER_URL, StringUtils.urlEncode(role) ) ); @@ -139,7 +143,7 @@ public void assignUserToRole(String user, String role) HttpMethod.POST, StringUtils.format( "%s/users/%s/roles/%s", - getAuthorizerURL(), + AUTHORIZER_URL, StringUtils.urlEncode(user), StringUtils.urlEncode(role) ) @@ -153,7 +157,7 @@ public void setPermissionsToRole(String role, List permissions) HttpMethod.POST, StringUtils.format( "%s/roles/%s/permissions/", - getAuthorizerURL(), + AUTHORIZER_URL, StringUtils.urlEncode(role) ) ); @@ -164,14 +168,4 @@ private void sendRequest(Function request) { clients.onLeaderCoordinator(request, null); } - - private String getAuthenticatorURL() - { - return "/druid-ext/basic-security/authentication/db/basic"; - } - - private String getAuthorizerURL() - { - return "/druid-ext/basic-security/authorization/db/basic"; - } } diff --git a/extensions-core/druid-basic-security/src/main/java/org/apache/druid/security/basic/authentication/db/cache/CoordinatorPollingBasicAuthenticatorCacheManager.java b/extensions-core/druid-basic-security/src/main/java/org/apache/druid/security/basic/authentication/db/cache/CoordinatorPollingBasicAuthenticatorCacheManager.java index 38158ece1eb9..6e2893b39c1d 100644 --- a/extensions-core/druid-basic-security/src/main/java/org/apache/druid/security/basic/authentication/db/cache/CoordinatorPollingBasicAuthenticatorCacheManager.java +++ b/extensions-core/druid-basic-security/src/main/java/org/apache/druid/security/basic/authentication/db/cache/CoordinatorPollingBasicAuthenticatorCacheManager.java @@ -24,7 +24,7 @@ import com.google.inject.Inject; import com.google.inject.Injector; import org.apache.commons.lang3.ArrayUtils; -import org.apache.druid.client.coordinator.CoordinatorServiceClient; +import org.apache.druid.client.coordinator.Coordinator; import org.apache.druid.concurrent.LifecycleLock; import org.apache.druid.guice.ManageLifecycle; import org.apache.druid.guice.annotations.Smile; @@ -40,6 +40,7 @@ import org.apache.druid.java.util.http.client.response.BytesFullResponseHandler; import org.apache.druid.java.util.http.client.response.BytesFullResponseHolder; import org.apache.druid.rpc.RequestBuilder; +import org.apache.druid.rpc.ServiceClient; import org.apache.druid.security.basic.BasicAuthCommonCacheConfig; import org.apache.druid.security.basic.BasicAuthUtils; import org.apache.druid.security.basic.authentication.BasicHTTPAuthenticator; @@ -73,7 +74,7 @@ public class CoordinatorPollingBasicAuthenticatorCacheManager implements BasicAu private final Injector injector; private final ObjectMapper objectMapper; private final LifecycleLock lifecycleLock = new LifecycleLock(); - private final CoordinatorServiceClient coordinatorClient; + private final ServiceClient coordinatorClient; private final BasicAuthCommonCacheConfig commonCacheConfig; private final ScheduledExecutorService exec; @@ -82,7 +83,7 @@ public CoordinatorPollingBasicAuthenticatorCacheManager( Injector injector, BasicAuthCommonCacheConfig commonCacheConfig, @Smile ObjectMapper objectMapper, - CoordinatorServiceClient coordinatorClient + @Coordinator ServiceClient coordinatorClient ) { this.exec = Execs.scheduledSingleThreaded("BasicAuthenticatorCacheManager-Exec--%d"); @@ -257,7 +258,7 @@ private Map tryFetchUserMapFromCoordinator(Strin HttpMethod.GET, StringUtils.format("/druid-ext/basic-security/authentication/db/%s/cachedSerializedUserMap", prefix) ); - BytesFullResponseHolder responseHolder = coordinatorClient.getServiceClient().request( + BytesFullResponseHolder responseHolder = coordinatorClient.request( req, new BytesFullResponseHandler() ); diff --git a/extensions-core/druid-basic-security/src/main/java/org/apache/druid/security/basic/authorization/db/cache/CoordinatorPollingBasicAuthorizerCacheManager.java b/extensions-core/druid-basic-security/src/main/java/org/apache/druid/security/basic/authorization/db/cache/CoordinatorPollingBasicAuthorizerCacheManager.java index 51bcf652eafd..91ccb334cd33 100644 --- a/extensions-core/druid-basic-security/src/main/java/org/apache/druid/security/basic/authorization/db/cache/CoordinatorPollingBasicAuthorizerCacheManager.java +++ b/extensions-core/druid-basic-security/src/main/java/org/apache/druid/security/basic/authorization/db/cache/CoordinatorPollingBasicAuthorizerCacheManager.java @@ -23,7 +23,7 @@ import com.google.common.base.Preconditions; import com.google.inject.Inject; import com.google.inject.Injector; -import org.apache.druid.client.coordinator.CoordinatorServiceClient; +import org.apache.druid.client.coordinator.Coordinator; import org.apache.druid.concurrent.LifecycleLock; import org.apache.druid.guice.ManageLifecycle; import org.apache.druid.guice.annotations.Smile; @@ -39,6 +39,7 @@ import org.apache.druid.java.util.http.client.response.BytesFullResponseHandler; import org.apache.druid.java.util.http.client.response.BytesFullResponseHolder; import org.apache.druid.rpc.RequestBuilder; +import org.apache.druid.rpc.ServiceClient; import org.apache.druid.security.basic.BasicAuthCommonCacheConfig; import org.apache.druid.security.basic.BasicAuthUtils; import org.apache.druid.security.basic.authorization.BasicRoleBasedAuthorizer; @@ -78,7 +79,7 @@ public class CoordinatorPollingBasicAuthorizerCacheManager implements BasicAutho private final Injector injector; private final ObjectMapper objectMapper; private final LifecycleLock lifecycleLock = new LifecycleLock(); - private final CoordinatorServiceClient coordinatorClient; + private final ServiceClient coordinatorClient; private final BasicAuthCommonCacheConfig commonCacheConfig; private final ScheduledExecutorService exec; @@ -87,7 +88,7 @@ public CoordinatorPollingBasicAuthorizerCacheManager( Injector injector, BasicAuthCommonCacheConfig commonCacheConfig, @Smile ObjectMapper objectMapper, - CoordinatorServiceClient coordinatorClient + @Coordinator ServiceClient coordinatorClient ) { this.exec = Execs.scheduledSingleThreaded("CoordinatorPollingBasicAuthorizerCacheManager-Exec--%d"); @@ -400,7 +401,7 @@ private UserAndRoleMap tryFetchUserMapsFromCoordinator( HttpMethod.GET, StringUtils.format("/druid-ext/basic-security/authorization/db/%s/cachedSerializedUserMap", prefix) ); - BytesFullResponseHolder responseHolder = coordinatorClient.getServiceClient().request( + BytesFullResponseHolder responseHolder = coordinatorClient.request( req, new BytesFullResponseHandler() ); @@ -424,7 +425,7 @@ private GroupMappingAndRoleMap tryFetchGroupMappingMapsFromCoordinator( HttpMethod.GET, StringUtils.format("/druid-ext/basic-security/authorization/db/%s/cachedSerializedGroupMappingMap", prefix) ); - BytesFullResponseHolder responseHolder = coordinatorClient.getServiceClient().request( + BytesFullResponseHolder responseHolder = coordinatorClient.request( req, new BytesFullResponseHandler() ); diff --git a/extensions-core/druid-basic-security/src/test/java/org/apache/druid/security/basic/authentication/db/cache/CoordinatorPollingBasicAuthenticatorCacheManagerTest.java b/extensions-core/druid-basic-security/src/test/java/org/apache/druid/security/basic/authentication/db/cache/CoordinatorPollingBasicAuthenticatorCacheManagerTest.java index 3b8fffddf6c4..63fca692623c 100644 --- a/extensions-core/druid-basic-security/src/test/java/org/apache/druid/security/basic/authentication/db/cache/CoordinatorPollingBasicAuthenticatorCacheManagerTest.java +++ b/extensions-core/druid-basic-security/src/test/java/org/apache/druid/security/basic/authentication/db/cache/CoordinatorPollingBasicAuthenticatorCacheManagerTest.java @@ -20,7 +20,6 @@ package org.apache.druid.security.basic.authentication.db.cache; import com.google.inject.Injector; -import org.apache.druid.client.coordinator.CoordinatorServiceClient; import org.apache.druid.java.util.common.StringUtils; import org.apache.druid.java.util.emitter.EmittingLogger; import org.apache.druid.java.util.metrics.StubServiceEmitter; @@ -63,9 +62,7 @@ public void test_stop_interruptsPollingThread() throws InterruptedException, IOE .andReturn(new AuthenticatorMapper(Map.of("test-basic-auth", authenticator))).once(); // Create a mock leader client and request - final CoordinatorServiceClient leaderClient = EasyMock.createStrictMock(CoordinatorServiceClient.class); final MockServiceClient serviceClient = new MockServiceClient(); - EasyMock.expect(leaderClient.getServiceClient()).andReturn(serviceClient).anyTimes(); // Return the first request immediately final String path = StringUtils.format( @@ -100,14 +97,14 @@ public ChannelBuffer getContent() } ); - EasyMock.replay(injector, leaderClient); + EasyMock.replay(injector); final int numRetries = 10; final CoordinatorPollingBasicAuthenticatorCacheManager manager = new CoordinatorPollingBasicAuthenticatorCacheManager( injector, new BasicAuthCommonCacheConfig(0L, 1L, temporaryFolder.newFolder().getAbsolutePath(), numRetries), TestHelper.JSON_MAPPER, - leaderClient + serviceClient ); // Start the manager and wait for a while to ensure that polling has started @@ -120,7 +117,7 @@ public ChannelBuffer getContent() Assert.assertTrue(isInterrupted.get()); - EasyMock.verify(injector, leaderClient); + EasyMock.verify(injector); } } diff --git a/extensions-core/druid-basic-security/src/test/java/org/apache/druid/security/basic/authorization/db/cache/CoordinatorPollingBasicAuthorizerCacheManagerTest.java b/extensions-core/druid-basic-security/src/test/java/org/apache/druid/security/basic/authorization/db/cache/CoordinatorPollingBasicAuthorizerCacheManagerTest.java index 3f76d56578ed..5823dc08bb4f 100644 --- a/extensions-core/druid-basic-security/src/test/java/org/apache/druid/security/basic/authorization/db/cache/CoordinatorPollingBasicAuthorizerCacheManagerTest.java +++ b/extensions-core/druid-basic-security/src/test/java/org/apache/druid/security/basic/authorization/db/cache/CoordinatorPollingBasicAuthorizerCacheManagerTest.java @@ -21,7 +21,6 @@ import com.fasterxml.jackson.databind.ObjectMapper; import com.google.inject.Injector; -import org.apache.druid.client.coordinator.CoordinatorServiceClient; import org.apache.druid.java.util.common.StringUtils; import org.apache.druid.java.util.common.jackson.JacksonUtils; import org.apache.druid.java.util.emitter.EmittingLogger; @@ -64,7 +63,6 @@ public class CoordinatorPollingBasicAuthorizerCacheManagerTest // Mocks private Injector injector; - private CoordinatorServiceClient leaderClient; private MockServiceClient serviceClient; private CoordinatorPollingBasicAuthorizerCacheManager manager; @@ -80,26 +78,24 @@ public void setup() throws IOException .andReturn(new AuthorizerMapper(Map.of(AUTHORIZER_NAME, authorizer))).once(); serviceClient = new MockServiceClient(); - leaderClient = EasyMock.createStrictMock(CoordinatorServiceClient.class); - EasyMock.expect(leaderClient.getServiceClient()).andReturn(serviceClient).anyTimes(); final int numRetries = 1; manager = new CoordinatorPollingBasicAuthorizerCacheManager( injector, new BasicAuthCommonCacheConfig(0L, 1L, temporaryFolder.newFolder().getAbsolutePath(), numRetries), MAPPER, - leaderClient + serviceClient ); } private void replayAll() { - EasyMock.replay(injector, leaderClient); + EasyMock.replay(injector); } private void verifyAll() { - EasyMock.verify(injector, leaderClient); + EasyMock.verify(injector); } @Test diff --git a/server/src/main/java/org/apache/druid/client/coordinator/CoordinatorServiceClient.java b/server/src/main/java/org/apache/druid/client/coordinator/CoordinatorServiceClient.java deleted file mode 100644 index 1e15ea445462..000000000000 --- a/server/src/main/java/org/apache/druid/client/coordinator/CoordinatorServiceClient.java +++ /dev/null @@ -1,45 +0,0 @@ -/* - * 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.client.coordinator; - -import org.apache.druid.rpc.ServiceClient; - -/** - * Wrapper over {@link ServiceClient} used to send requests to the Coordinator. - *

- * This client should be used to hit Coordinator APIs added by extensions. - * For core Coordinator APIs, use {@link CoordinatorClient} instead. - */ -public class CoordinatorServiceClient -{ - private final ServiceClient serviceClient; - - public CoordinatorServiceClient( - ServiceClient serviceClient - ) - { - this.serviceClient = serviceClient; - } - - public ServiceClient getServiceClient() - { - return serviceClient; - } -} diff --git a/server/src/main/java/org/apache/druid/rpc/guice/ServiceClientModule.java b/server/src/main/java/org/apache/druid/rpc/guice/ServiceClientModule.java index 1accdfb52ab6..203886f99fea 100644 --- a/server/src/main/java/org/apache/druid/rpc/guice/ServiceClientModule.java +++ b/server/src/main/java/org/apache/druid/rpc/guice/ServiceClientModule.java @@ -28,7 +28,6 @@ import org.apache.druid.client.coordinator.Coordinator; import org.apache.druid.client.coordinator.CoordinatorClient; import org.apache.druid.client.coordinator.CoordinatorClientImpl; -import org.apache.druid.client.coordinator.CoordinatorServiceClient; import org.apache.druid.client.indexing.IndexingService; import org.apache.druid.discovery.DruidNodeDiscoveryProvider; import org.apache.druid.discovery.NodeRole; @@ -133,20 +132,6 @@ public CoordinatorClient makeCoordinatorClient( return new CoordinatorClientImpl(serviceClient, jsonMapper); } - /** - * Creates a {@link CoordinatorServiceClient} used by extensions to send - * requests to the Coordinator. For core Coordinator APIs, - * {@link CoordinatorClient} should be used instead. - */ - @Provides - @LazySingleton - public static CoordinatorServiceClient createCoordinatorServiceClient( - @Coordinator final ServiceClient serviceClient - ) - { - return new CoordinatorServiceClient(serviceClient); - } - @Provides @ManageLifecycle @Broker diff --git a/server/src/test/java/org/apache/druid/rpc/guice/ServiceClientModuleTest.java b/server/src/test/java/org/apache/druid/rpc/guice/ServiceClientModuleTest.java index a084a1e53139..f039a023ad9f 100644 --- a/server/src/test/java/org/apache/druid/rpc/guice/ServiceClientModuleTest.java +++ b/server/src/test/java/org/apache/druid/rpc/guice/ServiceClientModuleTest.java @@ -22,15 +22,19 @@ import com.google.common.collect.ImmutableList; import com.google.inject.Guice; import com.google.inject.Injector; +import com.google.inject.Key; +import org.apache.druid.client.broker.Broker; import org.apache.druid.client.broker.BrokerClient; +import org.apache.druid.client.coordinator.Coordinator; import org.apache.druid.client.coordinator.CoordinatorClient; -import org.apache.druid.client.coordinator.CoordinatorServiceClient; +import org.apache.druid.client.indexing.IndexingService; import org.apache.druid.discovery.DruidNodeDiscoveryProvider; import org.apache.druid.guice.DruidGuiceExtensions; import org.apache.druid.guice.LifecycleModule; import org.apache.druid.guice.annotations.EscalatedGlobal; import org.apache.druid.jackson.JacksonModule; import org.apache.druid.java.util.http.client.HttpClient; +import org.apache.druid.rpc.ServiceClient; import org.apache.druid.rpc.ServiceClientFactory; import org.apache.druid.rpc.ServiceLocator; import org.apache.druid.rpc.indexing.OverlordClient; @@ -108,6 +112,18 @@ public void testGetBrokerClient() @Test public void testGetCoordinatorServiceClient() { - assertNotNull(injector.getInstance(CoordinatorServiceClient.class)); + assertNotNull(injector.getInstance(Key.get(ServiceClient.class, Coordinator.class))); + } + + @Test + public void testGetOverlordServiceClient() + { + assertNotNull(injector.getInstance(Key.get(ServiceClient.class, IndexingService.class))); + } + + @Test + public void testGetBrokerServiceClient() + { + assertNotNull(injector.getInstance(Key.get(ServiceClient.class, Broker.class))); } } diff --git a/services/src/test/java/org/apache/druid/testing/embedded/EmbeddedServiceClient.java b/services/src/test/java/org/apache/druid/testing/embedded/EmbeddedServiceClient.java index a9b9aafca95a..1f7d76a61981 100644 --- a/services/src/test/java/org/apache/druid/testing/embedded/EmbeddedServiceClient.java +++ b/services/src/test/java/org/apache/druid/testing/embedded/EmbeddedServiceClient.java @@ -52,6 +52,10 @@ /** * Contains {@link ServiceClient} objects to connect to various services in an * embedded test cluster. + * + * @see #onLeaderOverlord(Function) + * @see #onLeaderCoordinator(Function) + * @see #onAnyBroker(Function) */ public class EmbeddedServiceClient { @@ -114,6 +118,9 @@ public static EmbeddedServiceClient create(EmbeddedDruidCluster cluster, Escalat return new EmbeddedServiceClient(cluster, escalator); } + /** + * Stops the executor service used by this client. + */ public void stop() throws InterruptedException { clientConnectExec.shutdownNow(); From 8eb2ff454b4eb2951b4c7d66ffb05ff6e5b35b05 Mon Sep 17 00:00:00 2001 From: Kashif Faraz Date: Mon, 4 Aug 2025 14:21:28 +0530 Subject: [PATCH 04/10] Fix compilation and tests --- .../testing/embedded/compact/AutoCompactionTest.java | 12 ++++++------ .../embedded/server/HighAvailabilityTest.java | 4 ++-- 2 files changed, 8 insertions(+), 8 deletions(-) diff --git a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/compact/AutoCompactionTest.java b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/compact/AutoCompactionTest.java index 1c2f7fca78b2..dcf2b68efc7a 100644 --- a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/compact/AutoCompactionTest.java +++ b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/compact/AutoCompactionTest.java @@ -1671,7 +1671,7 @@ private void verifyScanResult(String field, String result) ); } - private void updateClusterConfig(ClusterCompactionConfig clusterConfig) throws Exception + private void updateClusterConfig(ClusterCompactionConfig clusterConfig) { compactionResource.updateClusterConfig(clusterConfig); LOG.info("Updated cluster config to [%s]", clusterConfig); @@ -1750,7 +1750,7 @@ private void submitCompactionConfig( AggregatorFactory[] metricsSpec, boolean dropExisting, CompactionEngine engine - ) throws Exception + ) { DataSourceCompactionConfig dataSourceCompactionConfig = InlineSchemaDataSourceCompactionConfig.builder() @@ -1800,7 +1800,7 @@ private void submitCompactionConfig( Assertions.assertEquals(foundDataSourceCompactionConfig.getSkipOffsetFromLatest(), skipOffsetFromLatest); } - private void deleteCompactionConfig() throws Exception + private void deleteCompactionConfig() { compactionResource.deleteDataSourceCompactionConfig(fullDatasourceName); @@ -1848,7 +1848,7 @@ private void forceTriggerAutoCompaction( } } - private void forceTriggerAutoCompaction(int numExpectedSegmentsAfterCompaction) throws Exception + private void forceTriggerAutoCompaction(int numExpectedSegmentsAfterCompaction) { compactionResource.forceTriggerAutoCompaction(); waitForCompactionToFinish(numExpectedSegmentsAfterCompaction); @@ -1924,7 +1924,7 @@ private void verifySegmentsCompactedDimensionSchema(List dimens } } - private void updateCompactionTaskSlot(double compactionTaskSlotRatio, int maxCompactionTaskSlots) throws Exception + private void updateCompactionTaskSlot(double compactionTaskSlotRatio, int maxCompactionTaskSlots) { final ClusterCompactionConfig oldConfig = compactionResource.getClusterConfig(); compactionResource.updateClusterConfig( @@ -1959,7 +1959,7 @@ private void getAndAssertCompactionStatus( long intervalCountAwaitingCompaction, long intervalCountCompacted, long intervalCountSkipped - ) throws Exception + ) { AutoCompactionSnapshot actualStatus = compactionResource.getCompactionStatus(fullDatasourceName); Assertions.assertNotNull(actualStatus); diff --git a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/server/HighAvailabilityTest.java b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/server/HighAvailabilityTest.java index 58a3a621ea25..9f2931c36cde 100644 --- a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/server/HighAvailabilityTest.java +++ b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/server/HighAvailabilityTest.java @@ -80,13 +80,13 @@ protected EmbeddedDruidCluster createCluster() return EmbeddedDruidCluster.withEmbeddedDerbyAndZookeeper() .useLatchableEmitter() + .addServer(router) .addServer(overlord1) .addServer(overlord2) .addServer(coordinator1) .addServer(coordinator2) .addServer(indexer) - .addServer(broker) - .addServer(router); + .addServer(broker); } @Test From 7803011a2b533c9a19013ff1a47260177ab50f1e Mon Sep 17 00:00:00 2001 From: Kashif Faraz Date: Mon, 4 Aug 2025 14:31:44 +0530 Subject: [PATCH 05/10] Minor fixes --- .../compact/CompactionResourceTestClient.java | 26 +++++++++---------- .../embedded/server/HighAvailabilityTest.java | 3 +++ 2 files changed, 16 insertions(+), 13 deletions(-) diff --git a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/compact/CompactionResourceTestClient.java b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/compact/CompactionResourceTestClient.java index 1f32b0ca62cc..4e9016b33283 100644 --- a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/compact/CompactionResourceTestClient.java +++ b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/compact/CompactionResourceTestClient.java @@ -47,11 +47,11 @@ public class CompactionResourceTestClient { private static final Logger log = new Logger(CompactionResourceTestClient.class); - private final EmbeddedServiceClient clients; + private final EmbeddedServiceClient client; CompactionResourceTestClient(EmbeddedDruidCluster cluster) { - this.clients = cluster.callApi().serviceClient(); + this.client = cluster.callApi().serviceClient(); } private String getCoordinatorURL() @@ -71,7 +71,7 @@ public void submitCompactionConfig(final DataSourceCompactionConfig dataSourceCo "%s/compaction/config/datasources/%s", getOverlordURL(), StringUtils.urlEncode(dataSource) ); - clients.onLeaderOverlord( + client.onLeaderOverlord( mapper -> new RequestBuilder(HttpMethod.POST, url).jsonContent( mapper, dataSourceCompactionConfig @@ -87,13 +87,13 @@ public void deleteDataSourceCompactionConfig(final String dataSource) "%s/compaction/config/datasources/%s", getOverlordURL(), StringUtils.urlEncode(dataSource) ); - clients.onLeaderOverlord(mapper -> new RequestBuilder(HttpMethod.DELETE, url), null); + client.onLeaderOverlord(mapper -> new RequestBuilder(HttpMethod.DELETE, url), null); } public List getAllCompactionConfigs() { String url = StringUtils.format("%s/compaction/config/datasources", getOverlordURL()); - final CompactionConfigsResponse payload = clients.onLeaderOverlord( + final CompactionConfigsResponse payload = client.onLeaderOverlord( mapper -> new RequestBuilder(HttpMethod.GET, url), new TypeReference<>() {} ); @@ -106,7 +106,7 @@ public DataSourceCompactionConfig getDataSourceCompactionConfig(String dataSourc "%s/compaction/config/datasources/%s", getOverlordURL(), StringUtils.urlEncode(dataSource) ); - return clients.onLeaderOverlord( + return client.onLeaderOverlord( mapper -> new RequestBuilder(HttpMethod.GET, url), new TypeReference<>() {} ); @@ -131,17 +131,17 @@ public void forceTriggerAutoCompaction() ); String url = StringUtils.format("%s/compaction/compact", getCoordinatorURL()); - clients.onLeaderCoordinator(mapper -> new RequestBuilder(HttpMethod.POST, url), null); + client.onLeaderCoordinator(mapper -> new RequestBuilder(HttpMethod.POST, url), null); } public void updateClusterConfig(ClusterCompactionConfig config) { - clients.onLeaderOverlord(o -> o.updateClusterCompactionConfig(config)); + client.onLeaderOverlord(o -> o.updateClusterCompactionConfig(config)); } public ClusterCompactionConfig getClusterConfig() { - return clients.onLeaderOverlord(OverlordClient::getClusterCompactionConfig); + return client.onLeaderOverlord(OverlordClient::getClusterCompactionConfig); } /** @@ -157,7 +157,7 @@ private void updateCompactionTaskSlot(Double compactionTaskSlotRatio, Integer ma StringUtils.urlEncode(compactionTaskSlotRatio.toString()), StringUtils.urlEncode(maxCompactionTaskSlots.toString()) ); - clients.onLeaderCoordinator(mapper -> new RequestBuilder(HttpMethod.POST, url), null); + client.onLeaderCoordinator(mapper -> new RequestBuilder(HttpMethod.POST, url), null); } public Map getCompactionProgress(String dataSource) @@ -167,7 +167,7 @@ public Map getCompactionProgress(String dataSource) getCoordinatorURL(), StringUtils.urlEncode(dataSource) ); - return clients.onLeaderCoordinator( + return client.onLeaderCoordinator( mapper -> new RequestBuilder(HttpMethod.GET, url), new TypeReference<>() {} ); @@ -182,7 +182,7 @@ public AutoCompactionSnapshot getCompactionStatus(String dataSource) ); try { - final CompactionStatusResponse latestSnapshots = clients.onLeaderCoordinator( + final CompactionStatusResponse latestSnapshots = client.onLeaderCoordinator( mapper -> new RequestBuilder(HttpMethod.GET, url), new TypeReference<>() {} ); @@ -198,7 +198,7 @@ public CompactionSimulateResult simulateRunOnCoordinator() final ClusterCompactionConfig clusterConfig = getClusterConfig(); final String url = StringUtils.format("%s/compaction/simulate", getCoordinatorURL()); - return clients.onLeaderCoordinator( + return client.onLeaderCoordinator( mapper -> new RequestBuilder(HttpMethod.POST, url).jsonContent(mapper, clusterConfig), new TypeReference<>() {} ); diff --git a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/server/HighAvailabilityTest.java b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/server/HighAvailabilityTest.java index 9f2931c36cde..e8d7382179d1 100644 --- a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/server/HighAvailabilityTest.java +++ b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/server/HighAvailabilityTest.java @@ -78,6 +78,9 @@ protected EmbeddedDruidCluster createCluster() .addProperty("druid.plaintextPort", "7081") .addProperty("druid.manager.segments.useIncrementalCache", "always"); + // Keep the Router first in the list to ensure that EmbeddedServiceClient + // does not use clients from the Overlord or Coordinator, which are stopped + // during the test and may cause failures due to ServiceClient being closed. return EmbeddedDruidCluster.withEmbeddedDerbyAndZookeeper() .useLatchableEmitter() .addServer(router) From eae15133b3bed65732c970948d5b8119687d56bc Mon Sep 17 00:00:00 2001 From: Kashif Faraz Date: Mon, 4 Aug 2025 14:36:56 +0530 Subject: [PATCH 06/10] Clean up --- .../druid/testing/embedded/EmbeddedDruidCluster.java | 3 ++- .../druid/testing/embedded/EmbeddedServiceClient.java | 8 ++++---- 2 files changed, 6 insertions(+), 5 deletions(-) diff --git a/services/src/test/java/org/apache/druid/testing/embedded/EmbeddedDruidCluster.java b/services/src/test/java/org/apache/druid/testing/embedded/EmbeddedDruidCluster.java index e2edad2749cd..dabceaff4b98 100644 --- a/services/src/test/java/org/apache/druid/testing/embedded/EmbeddedDruidCluster.java +++ b/services/src/test/java/org/apache/druid/testing/embedded/EmbeddedDruidCluster.java @@ -72,7 +72,7 @@ public class EmbeddedDruidCluster implements EmbeddedResource { private static final Logger log = new Logger(EmbeddedDruidCluster.class); - private final EmbeddedClusterApis clusterApis = new EmbeddedClusterApis(this); + private final EmbeddedClusterApis clusterApis; private final TestFolder testFolder = new TestFolder(); private final List> servers = new ArrayList<>(); @@ -86,6 +86,7 @@ public class EmbeddedDruidCluster implements EmbeddedResource private EmbeddedDruidCluster() { resources.add(testFolder); + clusterApis = new EmbeddedClusterApis(this); addExtension(RuntimeInfoModule.class); } diff --git a/services/src/test/java/org/apache/druid/testing/embedded/EmbeddedServiceClient.java b/services/src/test/java/org/apache/druid/testing/embedded/EmbeddedServiceClient.java index 1f7d76a61981..cf7322b1b8ff 100644 --- a/services/src/test/java/org/apache/druid/testing/embedded/EmbeddedServiceClient.java +++ b/services/src/test/java/org/apache/druid/testing/embedded/EmbeddedServiceClient.java @@ -50,8 +50,7 @@ import java.util.function.Function; /** - * Contains {@link ServiceClient} objects to connect to various services in an - * embedded test cluster. + * Client to make requests to various services in an embedded test cluster. * * @see #onLeaderOverlord(Function) * @see #onLeaderCoordinator(Function) @@ -77,15 +76,16 @@ private EmbeddedServiceClient(EmbeddedDruidCluster cluster, Escalator escalator) this.clientConnectExec = ScheduledExecutors.fixed(4, "ServiceClientFactory-%d"); this.responseHandler = StatusResponseHandler.getInstance(); + // If this server is stopped, the client becomes invalid final EmbeddedDruidServer anyServer = cluster.anyServer(); - final HttpClient escalatedClient = + final HttpClient escalatedHttpClient = escalator == null ? anyServer.bindings().escalatedHttpClient() : escalator.createEscalatedClient(anyServer.bindings().globalHttpClient()); // Create service clients - final ServiceClientFactory factory = new ServiceClientFactoryImpl(escalatedClient, clientConnectExec); + final ServiceClientFactory factory = new ServiceClientFactoryImpl(escalatedHttpClient, clientConnectExec); this.overlordServiceClient = module.makeServiceClientForOverlord( factory, anyServer.bindings().getInstance(ServiceLocator.class, IndexingService.class) From e70f828b72ce5b2be3c19e725ffa16547f7ad9df Mon Sep 17 00:00:00 2001 From: Kashif Faraz Date: Mon, 4 Aug 2025 22:29:03 +0530 Subject: [PATCH 07/10] Move test from ITCoordinatorOverlordProxyAuthTest to BasicAuthIndexingTest --- .../embedded/auth/BasicAuthIndexingTest.java | 21 ++++++++- .../CoordinatorResourceTestClient.java | 18 -------- .../ITCoordinatorOverlordProxyAuthTest.java | 44 ------------------- 3 files changed, 20 insertions(+), 63 deletions(-) delete mode 100644 integration-tests/src/test/java/org/apache/druid/tests/security/ITCoordinatorOverlordProxyAuthTest.java diff --git a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/auth/BasicAuthIndexingTest.java b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/auth/BasicAuthIndexingTest.java index 19f5d14ad6b8..71149fb6c138 100644 --- a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/auth/BasicAuthIndexingTest.java +++ b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/auth/BasicAuthIndexingTest.java @@ -19,9 +19,16 @@ package org.apache.druid.testing.embedded.auth; +import com.fasterxml.jackson.core.type.TypeReference; +import org.apache.druid.rpc.RequestBuilder; import org.apache.druid.security.basic.BasicSecurityDruidModule; import org.apache.druid.testing.embedded.EmbeddedDruidCluster; import org.apache.druid.testing.embedded.indexing.IndexTaskTest; +import org.jboss.netty.handler.codec.http.HttpMethod; +import org.junit.jupiter.api.Assertions; +import org.junit.jupiter.api.Test; + +import java.util.List; public class BasicAuthIndexingTest extends IndexTaskTest { @@ -41,6 +48,18 @@ public EmbeddedDruidCluster createCluster() .addCommonProperty("druid.escalator.type", "basic") .addCommonProperty("druid.escalator.internalClientPassword", "warlock") .addCommonProperty("druid.escalator.internalClientUsername", "druid_system") - .addCommonProperty("druid.escalator.authorizerName", "basic"); + .addCommonProperty("druid.escalator.authorizerName", "basic") + .addCommonProperty("druid.indexer.autoscale.doAutoscale", "true"); + } + + @Test + public void test_getScalingStats_redirectFromCoordinatorToOverlord() + { + final List response = cluster.callApi().serviceClient().onLeaderCoordinator( + mapper -> new RequestBuilder(HttpMethod.GET, "/druid/indexer/v1/scaling"), + new TypeReference<>() {} + ); + Assertions.assertNotNull(response); + Assertions.assertTrue(response.isEmpty()); } } 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 e82580cd312e..d7cfbfee6e6c 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 @@ -25,7 +25,6 @@ import com.google.common.net.HostAndPort; import com.google.inject.Inject; import org.apache.druid.java.util.common.ISE; -import org.apache.druid.java.util.common.RE; import org.apache.druid.java.util.common.StringUtils; import org.apache.druid.java.util.http.client.HttpClient; import org.apache.druid.java.util.http.client.Request; @@ -264,23 +263,6 @@ public void deleteSegmentsDataSource(String dataSource, Interval interval) } } - public HttpResponseStatus getProxiedOverlordScalingResponseStatus() - { - try { - StatusResponseHolder response = makeRequest( - HttpMethod.GET, - StringUtils.format( - "%s/druid/indexer/v1/scaling", - coordinator - ) - ); - return response.getStatus(); - } - catch (Exception e) { - throw new RE(e, "Unable to get scaling status from [%s]", coordinator); - } - } - public Map initializeLookups(String filePath) throws Exception { String url = StringUtils.format("%slookups/config", getCoordinatorURL()); diff --git a/integration-tests/src/test/java/org/apache/druid/tests/security/ITCoordinatorOverlordProxyAuthTest.java b/integration-tests/src/test/java/org/apache/druid/tests/security/ITCoordinatorOverlordProxyAuthTest.java deleted file mode 100644 index f6cc85ab1704..000000000000 --- a/integration-tests/src/test/java/org/apache/druid/tests/security/ITCoordinatorOverlordProxyAuthTest.java +++ /dev/null @@ -1,44 +0,0 @@ -/* - * 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.tests.security; - -import com.google.inject.Inject; -import org.apache.druid.testing.clients.CoordinatorResourceTestClient; -import org.apache.druid.testing.guice.DruidTestModuleFactory; -import org.apache.druid.tests.TestNGGroup; -import org.jboss.netty.handler.codec.http.HttpResponseStatus; -import org.testng.Assert; -import org.testng.annotations.Guice; -import org.testng.annotations.Test; - -@Test(groups = TestNGGroup.SECURITY) -@Guice(moduleFactory = DruidTestModuleFactory.class) -public class ITCoordinatorOverlordProxyAuthTest -{ - @Inject - CoordinatorResourceTestClient coordinatorClient; - - @Test - public void testProxyAuth() - { - HttpResponseStatus responseStatus = coordinatorClient.getProxiedOverlordScalingResponseStatus(); - Assert.assertEquals(responseStatus, HttpResponseStatus.OK); - } -} From 9b21d45007a554fe730ece5f244da0143074675c Mon Sep 17 00:00:00 2001 From: Kashif Faraz Date: Thu, 7 Aug 2025 15:17:43 +0530 Subject: [PATCH 08/10] Add EmbeddedBasicAuthResource --- .../embedded/auth/BasicAuthIndexingTest.java | 26 ++- ...AuthMsqTest.java => BasicAuthMSQTest.java} | 181 +----------------- .../auth/EmbeddedBasicAuthResource.java | 83 ++++++++ ...Directory.java => MSQExportDirectory.java} | 2 +- 4 files changed, 104 insertions(+), 188 deletions(-) rename embedded-tests/src/test/java/org/apache/druid/testing/embedded/auth/{BasicAuthMsqTest.java => BasicAuthMSQTest.java} (68%) create mode 100644 embedded-tests/src/test/java/org/apache/druid/testing/embedded/auth/EmbeddedBasicAuthResource.java rename embedded-tests/src/test/java/org/apache/druid/testing/embedded/msq/{MsqExportDirectory.java => MSQExportDirectory.java} (96%) diff --git a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/auth/BasicAuthIndexingTest.java b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/auth/BasicAuthIndexingTest.java index 71149fb6c138..91b36be7ea09 100644 --- a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/auth/BasicAuthIndexingTest.java +++ b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/auth/BasicAuthIndexingTest.java @@ -21,8 +21,8 @@ import com.fasterxml.jackson.core.type.TypeReference; import org.apache.druid.rpc.RequestBuilder; -import org.apache.druid.security.basic.BasicSecurityDruidModule; import org.apache.druid.testing.embedded.EmbeddedDruidCluster; +import org.apache.druid.testing.embedded.EmbeddedRouter; import org.apache.druid.testing.embedded.indexing.IndexTaskTest; import org.jboss.netty.handler.codec.http.HttpMethod; import org.junit.jupiter.api.Assertions; @@ -35,20 +35,16 @@ public class BasicAuthIndexingTest extends IndexTaskTest @Override public EmbeddedDruidCluster createCluster() { - return super - .createCluster() - .addExtension(BasicSecurityDruidModule.class) - .addCommonProperty("druid.auth.authenticatorChain", "[\"basic\"]") - .addCommonProperty("druid.auth.authenticator.basic.type", "basic") - .addCommonProperty("druid.auth.authenticator.basic.initialAdminPassword", "priest") - .addCommonProperty("druid.auth.authenticator.basic.initialInternalClientPassword", "warlock") - .addCommonProperty("druid.auth.authenticator.basic.authorizerName", "basic") - .addCommonProperty("druid.auth.authorizers", "[\"basic\"]") - .addCommonProperty("druid.auth.authorizer.basic.type", "basic") - .addCommonProperty("druid.escalator.type", "basic") - .addCommonProperty("druid.escalator.internalClientPassword", "warlock") - .addCommonProperty("druid.escalator.internalClientUsername", "druid_system") - .addCommonProperty("druid.escalator.authorizerName", "basic") + return EmbeddedDruidCluster + .withEmbeddedDerbyAndZookeeper() + .addResource(new EmbeddedBasicAuthResource()) + .useLatchableEmitter() + .addServer(coordinator) + .addServer(overlord) + .addServer(indexer) + .addServer(historical) + .addServer(broker) + .addServer(new EmbeddedRouter()) .addCommonProperty("druid.indexer.autoscale.doAutoscale", "true"); } diff --git a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/auth/BasicAuthMsqTest.java b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/auth/BasicAuthMSQTest.java similarity index 68% rename from embedded-tests/src/test/java/org/apache/druid/testing/embedded/auth/BasicAuthMsqTest.java rename to embedded-tests/src/test/java/org/apache/druid/testing/embedded/auth/BasicAuthMSQTest.java index 05e1ffe4516f..6e724ada65a8 100644 --- a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/auth/BasicAuthMsqTest.java +++ b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/auth/BasicAuthMSQTest.java @@ -20,12 +20,7 @@ package org.apache.druid.testing.embedded.auth; import com.google.common.collect.ImmutableList; -import org.apache.calcite.sql.type.SqlTypeName; -import org.apache.druid.common.utils.IdUtils; -import org.apache.druid.data.input.impl.JsonInputFormat; -import org.apache.druid.data.input.impl.LocalInputSource; import org.apache.druid.error.ExceptionMatcher; -import org.apache.druid.indexing.common.task.Task; import org.apache.druid.java.util.common.StringUtils; import org.apache.druid.metadata.DefaultPasswordProvider; import org.apache.druid.msq.guice.IndexerMemoryManagementModule; @@ -34,25 +29,12 @@ import org.apache.druid.msq.guice.MSQIndexingModule; import org.apache.druid.msq.guice.MSQSqlModule; import org.apache.druid.msq.guice.SqlTaskModule; -import org.apache.druid.msq.indexing.LegacyMSQSpec; -import org.apache.druid.msq.indexing.MSQControllerTask; -import org.apache.druid.msq.indexing.MSQTuningConfig; -import org.apache.druid.msq.indexing.destination.ExportMSQDestination; -import org.apache.druid.msq.kernel.WorkerAssignmentStrategy; -import org.apache.druid.query.Druids; import org.apache.druid.query.http.ClientSqlQuery; import org.apache.druid.query.http.SqlTaskStatus; -import org.apache.druid.security.basic.BasicSecurityDruidModule; import org.apache.druid.security.basic.authentication.BasicHTTPEscalator; -import org.apache.druid.segment.column.ColumnType; -import org.apache.druid.segment.column.RowSignature; import org.apache.druid.server.security.Action; import org.apache.druid.server.security.Resource; import org.apache.druid.server.security.ResourceAction; -import org.apache.druid.sql.calcite.external.ExternalDataSource; -import org.apache.druid.sql.calcite.planner.ColumnMapping; -import org.apache.druid.sql.calcite.planner.ColumnMappings; -import org.apache.druid.sql.http.ResultFormat; import org.apache.druid.storage.local.LocalFileExportStorageProvider; import org.apache.druid.storage.s3.output.S3ExportStorageProvider; import org.apache.druid.testing.embedded.EmbeddedBroker; @@ -63,7 +45,7 @@ import org.apache.druid.testing.embedded.EmbeddedServiceClient; import org.apache.druid.testing.embedded.indexing.Resources; import org.apache.druid.testing.embedded.junit5.EmbeddedClusterTestBase; -import org.apache.druid.testing.embedded.msq.MsqExportDirectory; +import org.apache.druid.testing.embedded.msq.MSQExportDirectory; import org.hamcrest.MatcherAssert; import org.junit.jupiter.api.AfterEach; import org.junit.jupiter.api.Assertions; @@ -75,24 +57,22 @@ import java.util.List; import java.util.Map; -public class BasicAuthMsqTest extends EmbeddedClusterTestBase +public class BasicAuthMSQTest extends EmbeddedClusterTestBase { public static final String USER_1 = "user1"; public static final String ROLE_1 = "role1"; public static final String USER_1_PASSWORD = "password1"; - // Time in ms to sleep after updating role permissions in each test. This intends to give the - // underlying test cluster enough time to sync permissions and be ready when test execution starts. - private static final int SYNC_SLEEP = 500; - private SecurityClient securityClient; private EmbeddedServiceClient userClient; - private final EmbeddedOverlord overlord = new EmbeddedOverlord(); + // Indexer with 2 slots, each with 150MB memory since minimum required memory + // computed for the required tests is 133MB private final EmbeddedIndexer indexer = new EmbeddedIndexer() - .setServerMemory(400_000_000) + .setServerMemory(300_000_000) .addProperty("druid.worker.capacity", "2"); - private final MsqExportDirectory exportDirectory = new MsqExportDirectory(); + private final EmbeddedOverlord overlord = new EmbeddedOverlord(); + private final MSQExportDirectory exportDirectory = new MSQExportDirectory(); @Override protected EmbeddedDruidCluster createCluster() @@ -101,12 +81,12 @@ protected EmbeddedDruidCluster createCluster() .withEmbeddedDerbyAndZookeeper() .useLatchableEmitter() .addResource(exportDirectory) + .addResource(new EmbeddedBasicAuthResource()) .addServer(new EmbeddedCoordinator()) .addServer(overlord) .addServer(indexer) .addServer(new EmbeddedBroker()) .addExtensions( - BasicSecurityDruidModule.class, MSQSqlModule.class, MSQIndexingModule.class, SqlTaskModule.class, @@ -114,18 +94,7 @@ protected EmbeddedDruidCluster createCluster() MSQExternalDataSourceModule.class, IndexerMemoryManagementModule.class ) - .addCommonProperty("druid.auth.basic.common.pollingPeriod", "100") - .addCommonProperty("druid.auth.authenticatorChain", "[\"basic\"]") - .addCommonProperty("druid.auth.authenticator.basic.type", "basic") - .addCommonProperty("druid.auth.authenticator.basic.initialAdminPassword", "priest") - .addCommonProperty("druid.auth.authenticator.basic.initialInternalClientPassword", "warlock") - .addCommonProperty("druid.auth.authenticator.basic.authorizerName", "basic") - .addCommonProperty("druid.auth.authorizers", "[\"basic\"]") - .addCommonProperty("druid.auth.authorizer.basic.type", "basic") - .addCommonProperty("druid.escalator.type", "basic") - .addCommonProperty("druid.escalator.internalClientPassword", "warlock") - .addCommonProperty("druid.escalator.internalClientUsername", "druid_system") - .addCommonProperty("druid.escalator.authorizerName", "basic"); + .addCommonProperty("druid.auth.basic.common.pollingPeriod", "100"); } @BeforeAll @@ -168,8 +137,6 @@ public void testIngestionWithoutPermissions() throws Exception List permissions = ImmutableList.of(); securityClient.setPermissionsToRole(ROLE_1, permissions); - waitForPermissionsToSync(); - String queryLocal = StringUtils.format( "INSERT INTO %s\n" @@ -224,8 +191,6 @@ public void testIngestionWithPermissions() throws Exception ); securityClient.setPermissionsToRole(ROLE_1, permissions); - waitForPermissionsToSync(); - String queryLocal = StringUtils.format( "INSERT INTO %s\n" @@ -287,8 +252,6 @@ public void testExportWithoutPermissions() throws InterruptedException ); securityClient.setPermissionsToRole(ROLE_1, permissions); - waitForPermissionsToSync(); - String exportQuery = StringUtils.format( "INSERT INTO extern(%s(exportPath => '%s'))\n" @@ -322,8 +285,6 @@ public void testExportWithPermissions() throws InterruptedException ); securityClient.setPermissionsToRole(ROLE_1, permissions); - waitForPermissionsToSync(); - String exportQuery = StringUtils.format( "INSERT INTO extern(%s(exportPath => '%s'))\n" @@ -349,130 +310,6 @@ public void testExportWithPermissions() throws InterruptedException cluster.callApi().waitForTaskToSucceed(taskStatus.getTaskId(), overlord); } - @Test - public void testExportTaskSubmitOverlordWithPermission() throws Exception - { - // No external write permissions for s3 - List permissions = ImmutableList.of( - new ResourceAction(new Resource(".*", "DATASOURCE"), Action.READ), - new ResourceAction(new Resource("EXTERNAL", "EXTERNAL"), Action.READ), - new ResourceAction(new Resource(LocalFileExportStorageProvider.TYPE_NAME, "EXTERNAL"), Action.WRITE), - new ResourceAction(new Resource("STATE", "STATE"), Action.READ), - new ResourceAction(new Resource(".*", "DATASOURCE"), Action.WRITE) - ); - securityClient.setPermissionsToRole(ROLE_1, permissions); - - waitForPermissionsToSync(); - - final String taskId = IdUtils.getRandomId(); - userClient.onLeaderOverlord(o -> o.runTask(taskId, createExportTask(taskId))); - cluster.callApi().waitForTaskToSucceed(taskId, overlord); - } - - @Test - public void testExportTaskSubmitOverlordWithoutPermission() throws Exception - { - // No external write permissions for s3 - List permissions = ImmutableList.of( - new ResourceAction(new Resource(".*", "DATASOURCE"), Action.READ), - new ResourceAction(new Resource("EXTERNAL", "EXTERNAL"), Action.READ), - new ResourceAction(new Resource(S3ExportStorageProvider.TYPE_NAME, "EXTERNAL"), Action.WRITE), - new ResourceAction(new Resource("STATE", "STATE"), Action.READ), - new ResourceAction(new Resource(".*", "DATASOURCE"), Action.WRITE) - ); - securityClient.setPermissionsToRole(ROLE_1, permissions); - - waitForPermissionsToSync(); - - final String taskId = IdUtils.getRandomId(); - MatcherAssert.assertThat( - Assertions.assertThrows( - Exception.class, - () -> userClient.onLeaderOverlord(o -> o.runTask(taskId, createExportTask(taskId))) - ), - ExceptionMatcher.of(Exception.class).expectMessageContains("403 Forbidden") - ); - } - - private Task createExportTask(String taskId) - { - final RowSignature rowSignature = RowSignature - .builder() - .add("timestamp", ColumnType.STRING) - .add("isRobot", ColumnType.STRING) - .add("diffUrl", ColumnType.STRING) - .add("added", ColumnType.LONG) - .add("countryIsoCode", ColumnType.STRING) - .add("regionName", ColumnType.STRING) - .add("channel", ColumnType.STRING) - .add("flags", ColumnType.STRING) - .add("delta", ColumnType.LONG) - .add("isUnpatrolled", ColumnType.STRING) - .add("isNew", ColumnType.STRING) - .add("deltaBucket", ColumnType.DOUBLE) - .add("isMinor", ColumnType.STRING) - .add("isAnonymous", ColumnType.STRING) - .add("deleted", ColumnType.LONG) - .add("cityName", ColumnType.STRING) - .add("metroCode", ColumnType.LONG) - .add("namespace", ColumnType.STRING) - .add("comment", ColumnType.STRING) - .add("page", ColumnType.STRING) - .add("commentLength", ColumnType.LONG) - .add("countryName", ColumnType.STRING) - .add("user", ColumnType.STRING) - .add("regionIsoCode", ColumnType.STRING) - .build(); - - return new MSQControllerTask( - taskId, - new LegacyMSQSpec( - new Druids.ScanQueryBuilder() - .columns("added", "delta", "page") - .dataSource( - new ExternalDataSource( - new LocalInputSource(null, null, List.of(Resources.DataFile.tinyWiki1Json()), null), - new JsonInputFormat(null, null, null, null, null), - rowSignature - ) - ) - .eternityInterval() - .context( - Map.of( - "scanSignature", - "[{\"name\":\"added\",\"type\":\"LONG\"},{\"name\":\"delta\",\"type\":\"LONG\"},{\"name\":\"page\",\"type\":\"STRING\"}]" - ) - ) - .build(), - new ColumnMappings( - List.of( - new ColumnMapping("page", "page"), - new ColumnMapping("added", "added"), - new ColumnMapping("delta", "delta") - ) - ), - new ExportMSQDestination( - new LocalFileExportStorageProvider(new File(exportDirectory.get(), dataSource).getAbsolutePath()), - ResultFormat.CSV - ), - WorkerAssignmentStrategy.MAX, - MSQTuningConfig.defaultConfig() - ), - null, - null, - null, - List.of(SqlTypeName.VARCHAR, SqlTypeName.BIGINT, SqlTypeName.BIGINT), - List.of(ColumnType.STRING, ColumnType.LONG, ColumnType.LONG), - null, - null - ); - } - - private void waitForPermissionsToSync() throws InterruptedException - { - Thread.sleep(SYNC_SLEEP); - } - private void verifySqlSubmitFailsWith403Forbidden(String sql) { MatcherAssert.assertThat( diff --git a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/auth/EmbeddedBasicAuthResource.java b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/auth/EmbeddedBasicAuthResource.java new file mode 100644 index 000000000000..3a54954287c3 --- /dev/null +++ b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/auth/EmbeddedBasicAuthResource.java @@ -0,0 +1,83 @@ +/* + * 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.testing.embedded.auth; + +import org.apache.druid.java.util.common.StringUtils; +import org.apache.druid.security.basic.BasicSecurityDruidModule; +import org.apache.druid.testing.embedded.EmbeddedDruidCluster; +import org.apache.druid.testing.embedded.EmbeddedResource; + +/** + * Resource to enable the basic auth extension in embedded tests. + */ +public class EmbeddedBasicAuthResource implements EmbeddedResource +{ + public static final String ADMIN_PASSWORD = "priest"; + public static final String SYSTEM_PASSWORD = "warlock"; + public static final String SYSTEM_USER = "druid_system"; + + private static final String AUTHORIZER_NAME = "basic"; + private static final String AUTHENTICATOR_NAME = "basic"; + + @Override + public void start() + { + // Do nothing + } + + @Override + public void onStarted(EmbeddedDruidCluster cluster) + { + cluster + .addExtension(BasicSecurityDruidModule.class) + .addCommonProperty("druid.auth.authenticatorChain", StringUtils.format("[\"%s\"]", AUTHENTICATOR_NAME)) + .addCommonProperty(authenticatorProp("type"), "basic") + .addCommonProperty(authenticatorProp("initialAdminPassword"), ADMIN_PASSWORD) + .addCommonProperty(authenticatorProp("initialInternalClientPassword"), SYSTEM_PASSWORD) + .addCommonProperty(authenticatorProp("authorizerName"), AUTHORIZER_NAME) + .addCommonProperty("druid.auth.authorizers", StringUtils.format("[\"%s\"]", AUTHORIZER_NAME)) + .addCommonProperty(authorizerProp("type"), "basic") + .addCommonProperty(escalatorProp("type"), "basic") + .addCommonProperty(escalatorProp("internalClientPassword"), SYSTEM_PASSWORD) + .addCommonProperty(escalatorProp("internalClientUsername"), SYSTEM_USER) + .addCommonProperty(escalatorProp("authorizerName"), AUTHORIZER_NAME); + } + + @Override + public void stop() + { + // Do nothing + } + + private String authenticatorProp(String name) + { + return StringUtils.format("druid.auth.authenticator.%s.%s", AUTHENTICATOR_NAME, name); + } + + private String authorizerProp(String name) + { + return StringUtils.format("druid.auth.authorizer.%s.%s", AUTHORIZER_NAME, name); + } + + private String escalatorProp(String name) + { + return StringUtils.format("druid.escalator.%s", name); + } +} diff --git a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/msq/MsqExportDirectory.java b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/msq/MSQExportDirectory.java similarity index 96% rename from embedded-tests/src/test/java/org/apache/druid/testing/embedded/msq/MsqExportDirectory.java rename to embedded-tests/src/test/java/org/apache/druid/testing/embedded/msq/MSQExportDirectory.java index 43fd18335c9e..f3c284ed32bd 100644 --- a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/msq/MsqExportDirectory.java +++ b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/msq/MSQExportDirectory.java @@ -27,7 +27,7 @@ /** * Embedded resource to set MSQ export directory. */ -public class MsqExportDirectory implements EmbeddedResource +public class MSQExportDirectory implements EmbeddedResource { private File directory; From 3181c1fa7413ac835b282f30a6cf980c40357da8 Mon Sep 17 00:00:00 2001 From: Kashif Faraz Date: Thu, 7 Aug 2025 15:18:49 +0530 Subject: [PATCH 09/10] Remove redundant throws clauses --- .../druid/testing/embedded/auth/BasicAuthMSQTest.java | 8 ++++---- 1 file changed, 4 insertions(+), 4 deletions(-) diff --git a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/auth/BasicAuthMSQTest.java b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/auth/BasicAuthMSQTest.java index 6e724ada65a8..f6da6c1109ef 100644 --- a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/auth/BasicAuthMSQTest.java +++ b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/auth/BasicAuthMSQTest.java @@ -132,7 +132,7 @@ public void tearDownRoles() } @Test - public void testIngestionWithoutPermissions() throws Exception + public void testIngestionWithoutPermissions() { List permissions = ImmutableList.of(); securityClient.setPermissionsToRole(ROLE_1, permissions); @@ -181,7 +181,7 @@ public void testIngestionWithoutPermissions() throws Exception } @Test - public void testIngestionWithPermissions() throws Exception + public void testIngestionWithPermissions() { List permissions = ImmutableList.of( new ResourceAction(new Resource(".*", "DATASOURCE"), Action.READ), @@ -240,7 +240,7 @@ public void testIngestionWithPermissions() throws Exception } @Test - public void testExportWithoutPermissions() throws InterruptedException + public void testExportWithoutPermissions() { // No external write permissions for s3 List permissions = ImmutableList.of( @@ -273,7 +273,7 @@ public void testExportWithoutPermissions() throws InterruptedException } @Test - public void testExportWithPermissions() throws InterruptedException + public void testExportWithPermissions() { // No external write permissions for s3 List permissions = ImmutableList.of( From ee26cd2b11297c2a846730eec6bcf5e6fa266466 Mon Sep 17 00:00:00 2001 From: Kashif Faraz Date: Thu, 7 Aug 2025 17:32:24 +0530 Subject: [PATCH 10/10] Minor cleanup to address strict compile issues --- .../embedded/compact/AutoCompactionTest.java | 35 ++++++++----------- 1 file changed, 14 insertions(+), 21 deletions(-) diff --git a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/compact/AutoCompactionTest.java b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/compact/AutoCompactionTest.java index dcf2b68efc7a..29991844f03a 100644 --- a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/compact/AutoCompactionTest.java +++ b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/compact/AutoCompactionTest.java @@ -225,7 +225,7 @@ public void setupClient() } @BeforeEach - public void resetCompactionTaskSlots() throws Exception + public void resetCompactionTaskSlots() { // Set compaction slot to 5 updateCompactionTaskSlot(0.5, 10); @@ -580,7 +580,6 @@ public void testAutoCompactionDutySubmitAndVerifyCompaction() throws Exception verifySegmentIntervals(intervalsBeforeCompaction); getAndAssertCompactionStatus( fullDatasourceName, - AutoCompactionSnapshot.ScheduleStatus.RUNNING, Matchers.equalTo(0L), Matchers.greaterThan(0L), Matchers.greaterThan(0L), @@ -598,7 +597,6 @@ public void testAutoCompactionDutySubmitAndVerifyCompaction() throws Exception verifySegmentIntervals(intervalsBeforeCompaction); getAndAssertCompactionStatus( fullDatasourceName, - AutoCompactionSnapshot.ScheduleStatus.RUNNING, Matchers.equalTo(0L), Matchers.greaterThan(0L), Matchers.equalTo(0L), @@ -642,7 +640,7 @@ public void testAutoCompactionDutyCanUpdateCompactionConfig(CompactionEngine eng LOG.info("Auto compaction test with hash partitioning"); final HashedPartitionsSpec hashedPartitionsSpec = new HashedPartitionsSpec(null, 3, null); - submitCompactionConfig(hashedPartitionsSpec, NO_SKIP_OFFSET, 1, null, null, null, null, false, engine); + submitCompactionConfig(hashedPartitionsSpec, NO_SKIP_OFFSET, null, null, null, null, false, engine); // 3 segments for both 2013-08-31 and 2013-09-01. (Note that numShards guarantees max shards but not exact // number of final shards, since some shards may end up empty.) forceTriggerAutoCompaction(6); @@ -669,7 +667,7 @@ public void testAutoCompactionDutyCanUpdateCompactionConfig(CompactionEngine eng false ); } - submitCompactionConfig(inputRangePartitionsSpec, NO_SKIP_OFFSET, 1, null, null, null, null, false, engine); + submitCompactionConfig(inputRangePartitionsSpec, NO_SKIP_OFFSET, null, null, null, null, false, engine); forceTriggerAutoCompaction(2); verifyQuery(INDEX_QUERIES_RESOURCE); verifySegmentsCompacted(expectedRangePartitionsSpec, 2); @@ -731,7 +729,6 @@ public void testAutoCompactionDutyCanUpdateTaskSlots() throws Exception verifySegmentIntervals(intervalsBeforeCompaction); getAndAssertCompactionStatus( fullDatasourceName, - AutoCompactionSnapshot.ScheduleStatus.RUNNING, Matchers.greaterThan(0L), Matchers.greaterThan(0L), Matchers.equalTo(0L), @@ -753,7 +750,6 @@ public void testAutoCompactionDutyCanUpdateTaskSlots() throws Exception verifySegmentIntervals(intervalsBeforeCompaction); getAndAssertCompactionStatus( fullDatasourceName, - AutoCompactionSnapshot.ScheduleStatus.RUNNING, Matchers.equalTo(0L), Matchers.greaterThan(0L), Matchers.equalTo(0L), @@ -1321,7 +1317,7 @@ public void testAutoCompactionDutyWithSegmentGranularityFinerAndNotAlignWithSegm List tasks = getCompleteTasksForDataSource(fullDatasourceName); TaskStatusPlus compactTask = null; for (TaskStatusPlus task : tasks) { - if (task.getType().equals("compact")) { + if ("compact".equals(task.getType())) { compactTask = task; } } @@ -1365,7 +1361,7 @@ public void testAutoCompactionDutyWithSegmentGranularityCoarserAndNotAlignWithSe List tasks = getCompleteTasksForDataSource(fullDatasourceName); TaskStatusPlus compactTask = null; for (TaskStatusPlus task : tasks) { - if (task.getType().equals("compact")) { + if ("compact".equals(task.getType())) { compactTask = task; } } @@ -1681,7 +1677,7 @@ private void submitCompactionConfig( Integer maxRowsPerSegment, Period skipOffsetFromLatest, CompactionEngine engine - ) throws Exception + ) { submitCompactionConfig(maxRowsPerSegment, skipOffsetFromLatest, null, engine); } @@ -1691,7 +1687,7 @@ private void submitCompactionConfig( Period skipOffsetFromLatest, UserCompactionTaskGranularityConfig granularitySpec, CompactionEngine engine - ) throws Exception + ) { submitCompactionConfig(maxRowsPerSegment, skipOffsetFromLatest, granularitySpec, false, engine); } @@ -1702,7 +1698,7 @@ private void submitCompactionConfig( UserCompactionTaskGranularityConfig granularitySpec, boolean dropExisting, CompactionEngine engine - ) throws Exception + ) { submitCompactionConfig( maxRowsPerSegment, @@ -1725,12 +1721,11 @@ private void submitCompactionConfig( AggregatorFactory[] metricsSpec, boolean dropExisting, CompactionEngine engine - ) throws Exception + ) { submitCompactionConfig( new DynamicPartitionsSpec(maxRowsPerSegment, null), skipOffsetFromLatest, - 1, granularitySpec, dimensionsSpec, transformSpec, @@ -1743,7 +1738,6 @@ private void submitCompactionConfig( private void submitCompactionConfig( PartitionsSpec partitionsSpec, Period skipOffsetFromLatest, - int maxNumConcurrentSubTasks, UserCompactionTaskGranularityConfig granularitySpec, UserCompactionTaskDimensionsConfig dimensionsSpec, CompactionTransformSpec transformSpec, @@ -1769,7 +1763,7 @@ private void submitCompactionConfig( null, null, null, - maxNumConcurrentSubTasks, + 1, null, null, null, @@ -1856,7 +1850,7 @@ private void forceTriggerAutoCompaction(int numExpectedSegmentsAfterCompaction) private void waitForCompactionToFinish(int numExpectedSegmentsAfterCompaction) { - final Set taskIds = getTaskIdsForState(null, dataSource); + final Set taskIds = getTaskIdsForState(dataSource); for (String taskId : taskIds) { cluster.callApi().waitForTaskToSucceed(taskId, overlord); } @@ -1949,7 +1943,6 @@ private void updateCompactionTaskSlot(double compactionTaskSlotRatio, int maxCom private void getAndAssertCompactionStatus( String fullDatasourceName, - AutoCompactionSnapshot.ScheduleStatus scheduleStatus, Matcher bytesAwaitingCompactionMatcher, Matcher bytesCompactedMatcher, Matcher bytesSkippedMatcher, @@ -1963,7 +1956,7 @@ private void getAndAssertCompactionStatus( { AutoCompactionSnapshot actualStatus = compactionResource.getCompactionStatus(fullDatasourceName); Assertions.assertNotNull(actualStatus); - Assertions.assertEquals(actualStatus.getScheduleStatus(), scheduleStatus); + Assertions.assertEquals(actualStatus.getScheduleStatus(), AutoCompactionSnapshot.ScheduleStatus.RUNNING); MatcherAssert.assertThat(actualStatus.getBytesAwaitingCompaction(), bytesAwaitingCompactionMatcher); MatcherAssert.assertThat(actualStatus.getBytesCompacted(), bytesCompactedMatcher); MatcherAssert.assertThat(actualStatus.getBytesSkipped(), bytesSkippedMatcher); @@ -1989,10 +1982,10 @@ private TaskPayloadResponse getTaskPayload(String taskId) return cluster.callApi().onLeaderOverlord(o -> o.taskPayload(taskId)); } - private Set getTaskIdsForState(String state, String dataSource) + private Set getTaskIdsForState(String dataSource) { return ImmutableList.copyOf( - (CloseableIterator) cluster.callApi().onLeaderOverlord(o -> o.taskStatuses(state, dataSource, 0)) + (CloseableIterator) cluster.callApi().onLeaderOverlord(o -> o.taskStatuses(null, dataSource, 0)) ).stream().map(TaskStatusPlus::getId).collect(Collectors.toSet()); } }