This is an automated email from the ASF dual-hosted git repository.

xvrl pushed a commit to branch kafka-3.1
in repository https://gitbox.apache.org/repos/asf/druid.git

commit 04c9475748b030df5e4a2b0abc8057e25243eb72
Author: Xavier Léauté <[email protected]>
AuthorDate: Wed Jan 26 15:00:22 2022 -0800

    fix use of private Kafka APIs and remove deprecated test rules
---
 .../lookup/KafkaLookupExtractorFactoryTest.java    | 46 +++++++++-------------
 .../query/lookup/TestKafkaExtractionCluster.java   |  4 +-
 .../druid/indexing/kafka/test/TestBroker.java      |  8 +---
 3 files changed, 21 insertions(+), 37 deletions(-)

diff --git 
a/extensions-core/kafka-extraction-namespace/src/test/java/org/apache/druid/query/lookup/KafkaLookupExtractorFactoryTest.java
 
b/extensions-core/kafka-extraction-namespace/src/test/java/org/apache/druid/query/lookup/KafkaLookupExtractorFactoryTest.java
index b5643ce..53ce0e4 100644
--- 
a/extensions-core/kafka-extraction-namespace/src/test/java/org/apache/druid/query/lookup/KafkaLookupExtractorFactoryTest.java
+++ 
b/extensions-core/kafka-extraction-namespace/src/test/java/org/apache/druid/query/lookup/KafkaLookupExtractorFactoryTest.java
@@ -27,18 +27,16 @@ import com.google.common.collect.ImmutableMap;
 import com.google.common.collect.Sets;
 import com.google.common.primitives.Bytes;
 import org.apache.druid.jackson.DefaultObjectMapper;
+import org.apache.druid.java.util.common.IAE;
 import org.apache.druid.server.lookup.namespace.cache.CacheHandler;
 import 
org.apache.druid.server.lookup.namespace.cache.NamespaceExtractionCacheManager;
 import org.apache.kafka.clients.consumer.Consumer;
 import org.apache.kafka.clients.consumer.MockConsumer;
 import org.apache.kafka.clients.consumer.OffsetResetStrategy;
 import org.easymock.EasyMock;
-import org.easymock.IAnswer;
 import org.junit.Assert;
 import org.junit.Before;
-import org.junit.Rule;
 import org.junit.Test;
-import org.junit.rules.ExpectedException;
 import org.junit.runner.RunWith;
 import org.powermock.api.easymock.PowerMock;
 import org.powermock.core.classloader.annotations.PowerMockIgnore;
@@ -77,10 +75,6 @@ public class KafkaLookupExtractorFactoryTest
       NamespaceExtractionCacheManager.class);
   private final CacheHandler cacheHandler = 
PowerMock.createStrictMock(CacheHandler.class);
 
-
-  @Rule
-  public ExpectedException expectedException = ExpectedException.none();
-
   @Before
   public void setUp()
   {
@@ -406,7 +400,6 @@ public class KafkaLookupExtractorFactoryTest
   @Test
   public void testStartFailsOnMissingConnect()
   {
-    expectedException.expectMessage("bootstrap.servers required property");
     EasyMock.expect(cacheManager.createCache())
             .andReturn(cacheHandler)
             .once();
@@ -418,7 +411,11 @@ public class KafkaLookupExtractorFactoryTest
         TOPIC,
         ImmutableMap.of()
     );
-    Assert.assertTrue(factory.start());
+    Assert.assertThrows(
+        "bootstrap.servers required property",
+        IAE.class,
+        () -> factory.start()
+    );
     Assert.assertTrue(factory.close());
     PowerMock.verify(cacheManager);
   }
@@ -426,8 +423,6 @@ public class KafkaLookupExtractorFactoryTest
   @Test
   public void testStartFailsOnGroupID()
   {
-    expectedException.expectMessage(
-        "Cannot set kafka property [group.id]. Property is randomly generated 
for you. Found");
     EasyMock.expect(cacheManager.createCache())
             .andReturn(cacheHandler)
             .once();
@@ -439,7 +434,11 @@ public class KafkaLookupExtractorFactoryTest
         TOPIC,
         ImmutableMap.of("group.id", "make me fail")
     );
-    Assert.assertTrue(factory.start());
+    Assert.assertThrows(
+        "Cannot set kafka property [group.id]. Property is randomly generated 
for you. Found",
+        IAE.class,
+        () -> factory.start()
+    );
     Assert.assertTrue(factory.close());
     PowerMock.verify(cacheManager);
   }
@@ -447,8 +446,6 @@ public class KafkaLookupExtractorFactoryTest
   @Test
   public void testStartFailsOnAutoOffset()
   {
-    expectedException.expectMessage(
-        "Cannot set kafka property [auto.offset.reset]. Property will be 
forced to [smallest]. Found ");
     EasyMock.expect(cacheManager.createCache())
             .andReturn(cacheHandler)
             .once();
@@ -461,7 +458,11 @@ public class KafkaLookupExtractorFactoryTest
         TOPIC,
         ImmutableMap.of("auto.offset.reset", "make me fail")
     );
-    Assert.assertTrue(factory.start());
+    Assert.assertThrows(
+        "Cannot set kafka property [auto.offset.reset]. Property will be 
forced to [smallest]. Found ",
+        IAE.class,
+        () -> factory.start()
+    );
     Assert.assertTrue(factory.close());
     PowerMock.verify(cacheManager);
   }
@@ -469,12 +470,11 @@ public class KafkaLookupExtractorFactoryTest
   @Test
   public void testFailsGetNotStarted()
   {
-    expectedException.expectMessage("Not started");
-    new KafkaLookupExtractorFactory(
+    Assert.assertThrows("Not started", NullPointerException.class, () -> new 
KafkaLookupExtractorFactory(
         cacheManager,
         TOPIC,
         DEFAULT_PROPERTIES
-    ).get();
+    ).get());
   }
 
   @Test
@@ -501,14 +501,4 @@ public class KafkaLookupExtractorFactoryTest
     Assert.assertEquals(connectTimeout, otherFactory.getConnectTimeout());
     Assert.assertEquals(injective, otherFactory.isInjective());
   }
-
-  private IAnswer<Boolean> getBlockingAnswer()
-  {
-    return () -> {
-      Thread.sleep(60000);
-      Assert.fail("Test failed to complete within 60000ms");
-
-      return false;
-    };
-  }
 }
diff --git 
a/extensions-core/kafka-extraction-namespace/src/test/java/org/apache/druid/query/lookup/TestKafkaExtractionCluster.java
 
b/extensions-core/kafka-extraction-namespace/src/test/java/org/apache/druid/query/lookup/TestKafkaExtractionCluster.java
index 318b713..1b74419 100644
--- 
a/extensions-core/kafka-extraction-namespace/src/test/java/org/apache/druid/query/lookup/TestKafkaExtractionCluster.java
+++ 
b/extensions-core/kafka-extraction-namespace/src/test/java/org/apache/druid/query/lookup/TestKafkaExtractionCluster.java
@@ -157,7 +157,7 @@ public class TestKafkaExtractionCluster
   private Map<String, String> getConsumerProperties()
   {
     final Map<String, String> props = new HashMap<>(KAFKA_PROPERTIES);
-    int port = 
kafkaServer.socketServer().config().advertisedListeners().apply(0).port();
+    int port = kafkaServer.advertisedListeners().apply(0).port();
     props.put("bootstrap.servers", StringUtils.format("127.0.0.1:%d", port));
     return props;
   }
@@ -201,7 +201,7 @@ public class TestKafkaExtractionCluster
   {
     final Properties kafkaProducerProperties = new Properties();
     kafkaProducerProperties.putAll(KAFKA_PROPERTIES);
-    int port = 
kafkaServer.socketServer().config().advertisedListeners().apply(0).port();
+    int port = kafkaServer.advertisedListeners().apply(0).port();
     kafkaProducerProperties.put("bootstrap.servers", 
StringUtils.format("127.0.0.1:%d", port));
     kafkaProducerProperties.put("key.serializer", 
ByteArraySerializer.class.getName());
     kafkaProducerProperties.put("value.serializer", 
ByteArraySerializer.class.getName());
diff --git 
a/extensions-core/kafka-indexing-service/src/test/java/org/apache/druid/indexing/kafka/test/TestBroker.java
 
b/extensions-core/kafka-indexing-service/src/test/java/org/apache/druid/indexing/kafka/test/TestBroker.java
index f028e4b..387bba8 100644
--- 
a/extensions-core/kafka-indexing-service/src/test/java/org/apache/druid/indexing/kafka/test/TestBroker.java
+++ 
b/extensions-core/kafka-indexing-service/src/test/java/org/apache/druid/indexing/kafka/test/TestBroker.java
@@ -26,7 +26,6 @@ import org.apache.druid.indexing.kafka.KafkaConsumerConfigs;
 import org.apache.druid.java.util.common.FileUtils;
 import org.apache.druid.java.util.common.StringUtils;
 import org.apache.kafka.clients.admin.Admin;
-import org.apache.kafka.clients.consumer.KafkaConsumer;
 import org.apache.kafka.clients.producer.KafkaProducer;
 import org.apache.kafka.common.serialization.ByteArraySerializer;
 import org.apache.kafka.common.utils.Time;
@@ -95,7 +94,7 @@ public class TestBroker implements Closeable
 
   public int getPort()
   {
-    return 
server.socketServer().config().advertisedListeners().apply(0).port();
+    return server.advertisedListeners().apply(0).port();
   }
 
   public KafkaProducer<byte[], byte[]> newProducer()
@@ -115,11 +114,6 @@ public class TestBroker implements Closeable
     return props;
   }
 
-  public KafkaConsumer<byte[], byte[]> newConsumer()
-  {
-    return new KafkaConsumer<>(consumerProperties());
-  }
-
   public Map<String, Object> producerProperties()
   {
     final Map<String, Object> props = new HashMap<>();

---------------------------------------------------------------------
To unsubscribe, e-mail: [email protected]
For additional commands, e-mail: [email protected]

Reply via email to