Skip to content

Commit 3f27c86

Browse files
fix: Change Kafka{Producer/Consumer} to fix the number of partitions instead of looking it up. (#322)
This prevents making too many queries to the number of partitions.
1 parent 21cf834 commit 3f27c86

File tree

8 files changed

+30
-72
lines changed

8 files changed

+30
-72
lines changed

pubsublite-kafka-shim/src/main/java/com/google/cloud/pubsublite/kafka/ConsumerSettings.java

Lines changed: 11 additions & 5 deletions
Original file line numberDiff line numberDiff line change
@@ -20,6 +20,7 @@
2020
import com.google.cloud.pubsublite.AdminClient;
2121
import com.google.cloud.pubsublite.AdminClientSettings;
2222
import com.google.cloud.pubsublite.CloudZone;
23+
import com.google.cloud.pubsublite.PartitionLookupUtils;
2324
import com.google.cloud.pubsublite.SubscriptionPath;
2425
import com.google.cloud.pubsublite.TopicPath;
2526
import com.google.cloud.pubsublite.cloudpubsub.FlowControlSettings;
@@ -67,12 +68,12 @@ public abstract static class Builder {
6768
}
6869

6970
public Consumer<byte[], byte[]> instantiate() throws StatusException {
70-
try {
71-
CloudZone zone = subscriptionPath().location();
72-
AdminClient adminClient =
73-
AdminClient.create(AdminClientSettings.newBuilder().setRegion(zone.region()).build());
71+
CloudZone zone = subscriptionPath().location();
72+
try (AdminClient adminClient =
73+
AdminClient.create(AdminClientSettings.newBuilder().setRegion(zone.region()).build())) {
7474
Subscription subscription = adminClient.getSubscription(subscriptionPath()).get();
7575
TopicPath topic = TopicPath.parse(subscription.getTopic());
76+
long partitionCount = PartitionLookupUtils.numPartitions(topic);
7677
AssignerFactory assignerFactory =
7778
receiver -> {
7879
AssignerBuilder.Builder builder = AssignerBuilder.newBuilder();
@@ -111,7 +112,12 @@ public Consumer<byte[], byte[]> instantiate() throws StatusException {
111112
CursorClient.create(CursorClientSettings.newBuilder().setRegion(zone.region()).build());
112113

113114
return new PubsubLiteConsumer(
114-
subscriptionPath(), topic, consumerFactory, assignerFactory, adminClient, cursorClient);
115+
subscriptionPath(),
116+
topic,
117+
partitionCount,
118+
consumerFactory,
119+
assignerFactory,
120+
cursorClient);
115121
} catch (Exception e) {
116122
throw ExtractStatus.toCanonical(e);
117123
}

pubsublite-kafka-shim/src/main/java/com/google/cloud/pubsublite/kafka/ProducerSettings.java

Lines changed: 3 additions & 7 deletions
Original file line numberDiff line numberDiff line change
@@ -17,9 +17,7 @@
1717
package com.google.cloud.pubsublite.kafka;
1818

1919
import com.google.auto.value.AutoValue;
20-
import com.google.cloud.pubsublite.AdminClient;
21-
import com.google.cloud.pubsublite.AdminClientSettings;
22-
import com.google.cloud.pubsublite.CloudZone;
20+
import com.google.cloud.pubsublite.PartitionLookupUtils;
2321
import com.google.cloud.pubsublite.TopicPath;
2422
import com.google.cloud.pubsublite.internal.wire.PubsubContext;
2523
import com.google.cloud.pubsublite.internal.wire.PubsubContext.Framework;
@@ -54,9 +52,7 @@ public Producer<byte[], byte[]> instantiate() throws StatusException {
5452
.setTopic(topicPath());
5553
RoutingPublisherBuilder.Builder routingBuilder =
5654
RoutingPublisherBuilder.newBuilder().setTopic(topicPath()).setPublisherBuilder(builder);
57-
CloudZone zone = topicPath().location();
58-
AdminClient adminClient =
59-
AdminClient.create(AdminClientSettings.newBuilder().setRegion(zone.region()).build());
60-
return new PubsubLiteProducer(routingBuilder.build(), adminClient, topicPath());
55+
return new PubsubLiteProducer(
56+
routingBuilder.build(), PartitionLookupUtils.numPartitions(topicPath()), topicPath());
6157
}
6258
}

pubsublite-kafka-shim/src/main/java/com/google/cloud/pubsublite/kafka/PubsubLiteConsumer.java

Lines changed: 6 additions & 12 deletions
Original file line numberDiff line numberDiff line change
@@ -20,7 +20,6 @@
2020

2121
import com.google.api.core.ApiFutureCallback;
2222
import com.google.api.core.ApiFutures;
23-
import com.google.cloud.pubsublite.AdminClient;
2423
import com.google.cloud.pubsublite.Offset;
2524
import com.google.cloud.pubsublite.Partition;
2625
import com.google.cloud.pubsublite.SubscriptionPath;
@@ -68,29 +67,29 @@
6867
* <p>This also filters methods that Pub/Sub Lite will not implement.
6968
*/
7069
class PubsubLiteConsumer implements Consumer<byte[], byte[]> {
71-
private static Duration INFINITE_DURATION = Duration.ofMillis(Long.MAX_VALUE);
72-
private static GoogleLogger logger = GoogleLogger.forEnclosingClass();
70+
private static final Duration INFINITE_DURATION = Duration.ofMillis(Long.MAX_VALUE);
71+
private static final GoogleLogger logger = GoogleLogger.forEnclosingClass();
7372
private final SubscriptionPath subscriptionPath;
7473
private final TopicPath topicPath;
74+
private final long partitionCount;
7575
private final ConsumerFactory consumerFactory;
7676
private final AssignerFactory assignerFactory;
77-
private final AdminClient adminClient;
7877
private final CursorClient cursorClient;
7978
private Optional<Assigner> assigner = Optional.empty();
8079
private Optional<SingleSubscriptionConsumer> consumer = Optional.empty();
8180

8281
PubsubLiteConsumer(
8382
SubscriptionPath subscriptionPath,
8483
TopicPath topicPath,
84+
long partitionCount,
8585
ConsumerFactory consumerFactory,
8686
AssignerFactory assignerFactory,
87-
AdminClient adminClient,
8887
CursorClient cursorClient) {
8988
this.subscriptionPath = subscriptionPath;
9089
this.topicPath = topicPath;
90+
this.partitionCount = partitionCount;
9191
this.consumerFactory = consumerFactory;
9292
this.assignerFactory = assignerFactory;
93-
this.adminClient = adminClient;
9493
this.cursorClient = cursorClient;
9594
}
9695

@@ -443,7 +442,7 @@ public List<PartitionInfo> partitionsFor(String s) {
443442
@Override
444443
public List<PartitionInfo> partitionsFor(String topic, Duration timeout) {
445444
checkTopic(topic);
446-
return SharedBehavior.partitionsFor(adminClient, topicPath, timeout);
445+
return SharedBehavior.partitionsFor(partitionCount, topicPath);
447446
}
448447

449448
@Override
@@ -509,11 +508,6 @@ public void close(long l, TimeUnit timeUnit) {
509508

510509
@Override
511510
public void close(Duration timeout) {
512-
try {
513-
adminClient.close();
514-
} catch (Exception e) {
515-
logger.atSevere().withCause(e).log("Error closing admin client during Consumer shutdown.");
516-
}
517511
try {
518512
cursorClient.close();
519513
} catch (Exception e) {

pubsublite-kafka-shim/src/main/java/com/google/cloud/pubsublite/kafka/PubsubLiteProducer.java

Lines changed: 4 additions & 11 deletions
Original file line numberDiff line numberDiff line change
@@ -24,7 +24,6 @@
2424
import com.google.api.core.ApiFutures;
2525
import com.google.api.core.ApiService.Listener;
2626
import com.google.api.core.ApiService.State;
27-
import com.google.cloud.pubsublite.AdminClient;
2827
import com.google.cloud.pubsublite.PublishMetadata;
2928
import com.google.cloud.pubsublite.TopicPath;
3029
import com.google.cloud.pubsublite.internal.ExtractStatus;
@@ -52,21 +51,20 @@
5251
import org.apache.kafka.common.errors.UnsupportedVersionException;
5352

5453
class PubsubLiteProducer implements Producer<byte[], byte[]> {
55-
private static Duration INFINITE_DURATION = Duration.ofMillis(Long.MAX_VALUE);
5654
private static final UnsupportedVersionException NO_TRANSACTIONS_EXCEPTION =
5755
new UnsupportedVersionException(
5856
"Pub/Sub Lite is a non-transactional system and does not support producer transactions.");
5957
private static final GoogleLogger logger = GoogleLogger.forEnclosingClass();
6058

6159
private final Publisher<PublishMetadata> publisher;
62-
private final AdminClient adminClient;
6360
private final TopicPath topicPath;
61+
private final long partitionCount;
6462

6563
PubsubLiteProducer(
66-
Publisher<PublishMetadata> publisher, AdminClient adminClient, TopicPath topicPath) {
64+
Publisher<PublishMetadata> publisher, long partitionCount, TopicPath topicPath) {
6765
this.publisher = publisher;
68-
this.adminClient = adminClient;
6966
this.topicPath = topicPath;
67+
this.partitionCount = partitionCount;
7068
this.publisher.addListener(
7169
new Listener() {
7270
@Override
@@ -177,7 +175,7 @@ public void flush() {
177175
@Override
178176
public List<PartitionInfo> partitionsFor(String s) {
179177
checkTopic(s);
180-
return SharedBehavior.partitionsFor(adminClient, topicPath, INFINITE_DURATION);
178+
return SharedBehavior.partitionsFor(partitionCount, topicPath);
181179
}
182180

183181
@Override
@@ -192,11 +190,6 @@ public void close() {
192190

193191
@Override
194192
public void close(Duration duration) {
195-
try {
196-
adminClient.close();
197-
} catch (Exception e) {
198-
logger.atWarning().withCause(e).log("Failed to close admin client.");
199-
}
200193
try {
201194
publisher.stopAsync().awaitTerminated(duration.toMillis(), MILLISECONDS);
202195
} catch (TimeoutException e) {

pubsublite-kafka-shim/src/main/java/com/google/cloud/pubsublite/kafka/SharedBehavior.java

Lines changed: 2 additions & 8 deletions
Original file line numberDiff line numberDiff line change
@@ -18,13 +18,10 @@
1818

1919
import static com.google.cloud.pubsublite.kafka.KafkaExceptionUtils.toKafka;
2020

21-
import com.google.cloud.pubsublite.AdminClient;
2221
import com.google.cloud.pubsublite.Partition;
2322
import com.google.cloud.pubsublite.TopicPath;
2423
import com.google.common.collect.ImmutableList;
25-
import java.time.Duration;
2624
import java.util.List;
27-
import java.util.concurrent.TimeUnit;
2825
import org.apache.kafka.common.PartitionInfo;
2926

3027
/** Shared behavior for producer and consumer. */
@@ -40,13 +37,10 @@ static PartitionInfo toPartitionInfo(TopicPath topic, Partition partition) {
4037
PubsubLiteNode.NODES);
4138
}
4239

43-
static List<PartitionInfo> partitionsFor(
44-
AdminClient adminClient, TopicPath topic, Duration timeout) {
40+
static List<PartitionInfo> partitionsFor(long partitionCount, TopicPath topic) {
4541
try {
46-
long count =
47-
adminClient.getTopicPartitionCount(topic).get(timeout.toMillis(), TimeUnit.MILLISECONDS);
4842
ImmutableList.Builder<PartitionInfo> result = ImmutableList.builder();
49-
for (int i = 0; i < count; ++i) {
43+
for (int i = 0; i < partitionCount; ++i) {
5044
result.add(toPartitionInfo(topic, Partition.of(i)));
5145
}
5246
return result.build();

pubsublite-kafka-shim/src/test/java/com/google/cloud/pubsublite/kafka/PubsubLiteConsumerTest.java

Lines changed: 1 addition & 3 deletions
Original file line numberDiff line numberDiff line change
@@ -30,7 +30,6 @@
3030

3131
import com.google.api.core.ApiFutures;
3232
import com.google.api.core.SettableApiFuture;
33-
import com.google.cloud.pubsublite.AdminClient;
3433
import com.google.cloud.pubsublite.CloudZone;
3534
import com.google.cloud.pubsublite.Offset;
3635
import com.google.cloud.pubsublite.Partition;
@@ -98,7 +97,6 @@ private static <T> T example(Class<T> klass) {
9897

9998
@Mock ConsumerFactory consumerFactory;
10099
@Mock AssignerFactory assignerFactory;
101-
@Mock AdminClient adminClient;
102100
@Mock CursorClient cursorClient;
103101

104102
@Mock Assigner assigner;
@@ -113,9 +111,9 @@ public void setUp() {
113111
new PubsubLiteConsumer(
114112
example(SubscriptionPath.class),
115113
example(TopicPath.class),
114+
3,
116115
consumerFactory,
117116
assignerFactory,
118-
adminClient,
119117
cursorClient);
120118
when(consumerFactory.newConsumer()).thenReturn(underlying);
121119
}

pubsublite-kafka-shim/src/test/java/com/google/cloud/pubsublite/kafka/PubsubLiteProducerTest.java

Lines changed: 1 addition & 6 deletions
Original file line numberDiff line numberDiff line change
@@ -24,7 +24,6 @@
2424
import static org.mockito.Mockito.when;
2525

2626
import com.google.api.core.SettableApiFuture;
27-
import com.google.cloud.pubsublite.AdminClient;
2827
import com.google.cloud.pubsublite.Message;
2928
import com.google.cloud.pubsublite.Offset;
3029
import com.google.cloud.pubsublite.Partition;
@@ -49,7 +48,6 @@
4948
import org.junit.Test;
5049
import org.junit.runner.RunWith;
5150
import org.junit.runners.JUnit4;
52-
import org.mockito.Mock;
5351
import org.mockito.MockitoAnnotations;
5452
import org.mockito.Spy;
5553

@@ -68,14 +66,12 @@ abstract static class FakePublisher extends FakeApiService
6866

6967
@Spy FakePublisher underlying;
7068

71-
@Mock AdminClient adminClient;
72-
7369
Producer<byte[], byte[]> producer;
7470

7571
@Before
7672
public void setUp() {
7773
MockitoAnnotations.initMocks(this);
78-
producer = new PubsubLiteProducer(underlying, adminClient, example(TopicPath.class));
74+
producer = new PubsubLiteProducer(underlying, 3, example(TopicPath.class));
7975
verify(underlying).startAsync();
8076
verify(underlying).awaitRunning();
8177
}
@@ -212,7 +208,6 @@ public void flush() throws Exception {
212208
@Test
213209
public void close() throws Exception {
214210
producer.close();
215-
verify(adminClient).close();
216211
verify(underlying).stopAsync();
217212
verify(underlying).awaitTerminated(Long.MAX_VALUE, TimeUnit.MILLISECONDS);
218213
}

pubsublite-kafka-shim/src/test/java/com/google/cloud/pubsublite/kafka/SharedBehaviorTest.java

Lines changed: 2 additions & 20 deletions
Original file line numberDiff line numberDiff line change
@@ -18,37 +18,19 @@
1818

1919
import static com.google.cloud.pubsublite.internal.testing.UnitTestExamples.example;
2020
import static com.google.common.truth.Truth.assertThat;
21-
import static org.mockito.Mockito.when;
22-
import static org.mockito.MockitoAnnotations.initMocks;
2321

24-
import com.google.api.core.ApiFuture;
25-
import com.google.api.core.ApiFutures;
26-
import com.google.cloud.pubsublite.AdminClient;
2722
import com.google.cloud.pubsublite.TopicPath;
28-
import java.time.Duration;
2923
import java.util.List;
3024
import org.apache.kafka.common.PartitionInfo;
31-
import org.junit.Before;
3225
import org.junit.Test;
3326
import org.junit.runner.RunWith;
3427
import org.junit.runners.JUnit4;
35-
import org.mockito.Mock;
3628

3729
@RunWith(JUnit4.class)
3830
public class SharedBehaviorTest {
39-
@Mock AdminClient admin;
40-
41-
@Before
42-
public void setUp() {
43-
initMocks(this);
44-
}
45-
4631
@Test
47-
public void partitionsForSuccess() throws Exception {
48-
ApiFuture<Long> future = ApiFutures.immediateFuture(2L);
49-
when(admin.getTopicPartitionCount(example(TopicPath.class))).thenReturn(future);
50-
List<PartitionInfo> result =
51-
SharedBehavior.partitionsFor(admin, example(TopicPath.class), Duration.ofDays(1));
32+
public void partitionsForSuccess() {
33+
List<PartitionInfo> result = SharedBehavior.partitionsFor(2, example(TopicPath.class));
5234
assertThat(result.size()).isEqualTo(2);
5335
assertThat(result.get(0).topic()).isEqualTo(example(TopicPath.class).toString());
5436
assertThat(result.get(0).partition()).isEqualTo(0);

0 commit comments

Comments
 (0)