diff --git a/ksql-engine/src/main/java/io/confluent/ksql/services/ConfiguredKafkaClientSupplier.java b/ksql-engine/src/main/java/io/confluent/ksql/services/ConfiguredKafkaClientSupplier.java index 6c4811fd2c41..cde6956e8430 100644 --- a/ksql-engine/src/main/java/io/confluent/ksql/services/ConfiguredKafkaClientSupplier.java +++ b/ksql-engine/src/main/java/io/confluent/ksql/services/ConfiguredKafkaClientSupplier.java @@ -19,7 +19,7 @@ import java.util.HashMap; import java.util.Map; import java.util.Objects; -import org.apache.kafka.clients.admin.AdminClient; +import org.apache.kafka.clients.admin.Admin; import org.apache.kafka.clients.consumer.Consumer; import org.apache.kafka.clients.producer.Producer; import org.apache.kafka.streams.KafkaClientSupplier; @@ -49,8 +49,8 @@ public Map injectSupplierProperties(final Map co } @Override - public AdminClient getAdminClient(final Map config) { - return defaultSupplier.getAdminClient(injectSupplierProperties(config)); + public Admin getAdmin(final Map config) { + return defaultSupplier.getAdmin(injectSupplierProperties(config)); } @Override diff --git a/ksql-engine/src/main/java/io/confluent/ksql/services/SandboxedKafkaClientSupplier.java b/ksql-engine/src/main/java/io/confluent/ksql/services/SandboxedKafkaClientSupplier.java index cae07f003feb..8fc079119cf7 100644 --- a/ksql-engine/src/main/java/io/confluent/ksql/services/SandboxedKafkaClientSupplier.java +++ b/ksql-engine/src/main/java/io/confluent/ksql/services/SandboxedKafkaClientSupplier.java @@ -16,7 +16,7 @@ package io.confluent.ksql.services; import java.util.Map; -import org.apache.kafka.clients.admin.AdminClient; +import org.apache.kafka.clients.admin.Admin; import org.apache.kafka.clients.consumer.Consumer; import org.apache.kafka.clients.producer.Producer; import org.apache.kafka.streams.KafkaClientSupplier; @@ -35,7 +35,7 @@ class SandboxedKafkaClientSupplier implements KafkaClientSupplier { } @Override - public AdminClient getAdminClient(final Map config) { + public Admin getAdmin(final Map config) { return new SandboxedAdminClient(); } diff --git a/ksql-engine/src/main/java/io/confluent/ksql/services/ServiceContextFactory.java b/ksql-engine/src/main/java/io/confluent/ksql/services/ServiceContextFactory.java index c38789360044..148981e2d3b5 100644 --- a/ksql-engine/src/main/java/io/confluent/ksql/services/ServiceContextFactory.java +++ b/ksql-engine/src/main/java/io/confluent/ksql/services/ServiceContextFactory.java @@ -40,7 +40,7 @@ public static ServiceContext create( final KafkaClientSupplier kafkaClientSupplier, final Supplier srClientFactory ) { - final Admin adminClient = kafkaClientSupplier.getAdminClient( + final Admin adminClient = kafkaClientSupplier.getAdmin( ksqlConfig.getKsqlAdminClientConfigProps() ); diff --git a/ksql-engine/src/test/java/io/confluent/ksql/KsqlContextTestUtil.java b/ksql-engine/src/test/java/io/confluent/ksql/KsqlContextTestUtil.java index 1cdc6c36a3aa..c7fd85b68d42 100644 --- a/ksql-engine/src/test/java/io/confluent/ksql/KsqlContextTestUtil.java +++ b/ksql-engine/src/test/java/io/confluent/ksql/KsqlContextTestUtil.java @@ -46,7 +46,7 @@ public static KsqlContext create( final KafkaClientSupplier clientSupplier = new DefaultKafkaClientSupplier(); final Admin adminClient = clientSupplier - .getAdminClient(ksqlConfig.getKsqlAdminClientConfigProps()); + .getAdmin(ksqlConfig.getKsqlAdminClientConfigProps()); final KafkaTopicClient kafkaTopicClient = new KafkaTopicClientImpl(adminClient); diff --git a/ksql-engine/src/test/java/io/confluent/ksql/services/SandboxedKafkaClientSupplierTest.java b/ksql-engine/src/test/java/io/confluent/ksql/services/SandboxedKafkaClientSupplierTest.java index 474417a3d52f..c9b53aa0a1ee 100644 --- a/ksql-engine/src/test/java/io/confluent/ksql/services/SandboxedKafkaClientSupplierTest.java +++ b/ksql-engine/src/test/java/io/confluent/ksql/services/SandboxedKafkaClientSupplierTest.java @@ -46,7 +46,7 @@ public static class UnsupportedMethods { @Parameterized.Parameters(name = "{0}") public static Collection> getMethodsToTest() { return TestMethods.builder(SandboxedKafkaClientSupplier.class) - .ignore("getAdminClient", Map.class) + .ignore("getAdmin", Map.class) .ignore("getProducer", Map.class) .ignore("getConsumer", Map.class) .ignore("getRestoreConsumer", Map.class) @@ -83,7 +83,7 @@ public void setUp() { @Test public void shouldReturnTryAdminClient() { - assertThat(sandboxedKafkaClientSupplier.getAdminClient(config), + assertThat(sandboxedKafkaClientSupplier.getAdmin(config), is(instanceOf(SandboxedAdminClient.class))); } diff --git a/ksql-engine/src/test/java/io/confluent/ksql/services/TestServiceContext.java b/ksql-engine/src/test/java/io/confluent/ksql/services/TestServiceContext.java index e19ddb755248..7c94d387974b 100644 --- a/ksql-engine/src/test/java/io/confluent/ksql/services/TestServiceContext.java +++ b/ksql-engine/src/test/java/io/confluent/ksql/services/TestServiceContext.java @@ -60,7 +60,7 @@ public static ServiceContext create( ) { return create( new FakeKafkaClientSupplier(), - new FakeKafkaClientSupplier().getAdminClient(Collections.emptyMap()), + new FakeKafkaClientSupplier().getAdmin(Collections.emptyMap()), topicClient, srClientFactory, new DefaultConnectClient("http://localhost:8083") @@ -73,7 +73,7 @@ public static ServiceContext create( ) { final DefaultKafkaClientSupplier kafkaClientSupplier = new DefaultKafkaClientSupplier(); final Admin adminClient = kafkaClientSupplier - .getAdminClient(ksqlConfig.getKsqlAdminClientConfigProps()); + .getAdmin(ksqlConfig.getKsqlAdminClientConfigProps()); return create( kafkaClientSupplier, diff --git a/ksql-engine/src/test/java/io/confluent/ksql/util/FakeKafkaClientSupplier.java b/ksql-engine/src/test/java/io/confluent/ksql/util/FakeKafkaClientSupplier.java index 5f7438115f94..48fc7ca30eb9 100644 --- a/ksql-engine/src/test/java/io/confluent/ksql/util/FakeKafkaClientSupplier.java +++ b/ksql-engine/src/test/java/io/confluent/ksql/util/FakeKafkaClientSupplier.java @@ -17,7 +17,7 @@ import java.util.Collections; import java.util.Map; -import org.apache.kafka.clients.admin.AdminClient; +import org.apache.kafka.clients.admin.Admin; import org.apache.kafka.clients.admin.MockAdminClient; import org.apache.kafka.clients.consumer.Consumer; import org.apache.kafka.clients.consumer.MockConsumer; @@ -30,7 +30,7 @@ public class FakeKafkaClientSupplier implements KafkaClientSupplier { @Override - public AdminClient getAdminClient(final Map config) { + public Admin getAdmin(final Map config) { final Node node = new Node(1, "localhost", 1234); return new MockAdminClient(Collections.singletonList(node), node); }