You are viewing a plain text version of this content. The canonical link for it is here.
Posted to jira@kafka.apache.org by GitBox <gi...@apache.org> on 2020/09/20 03:08:26 UTC

[GitHub] [kafka] guozhangwang commented on a change in pull request #9060: KAFKA-9274: Remove `retries` from InternalTopicManager

guozhangwang commented on a change in pull request #9060:
URL: https://github.com/apache/kafka/pull/9060#discussion_r491645780



##########
File path: streams/src/main/java/org/apache/kafka/streams/processor/internals/InternalTopicManager.java
##########
@@ -45,33 +49,38 @@
         "Please report at https://issues.apache.org/jira/projects/KAFKA or dev-mailing list (https://kafka.apache.org/contact).";
 
     private final Logger log;
-    private final long windowChangeLogAdditionalRetention;
-    private final Map<String, String> defaultTopicConfigs = new HashMap<>();
 
-    private final short replicationFactor;
+    private final Time time;
     private final Admin adminClient;
 
-    private final int retries;
+    private final short replicationFactor;
+    private final long windowChangeLogAdditionalRetention;
     private final long retryBackOffMs;
+    private final long retryTimeoutMs;
+
+    private final Map<String, String> defaultTopicConfigs = new HashMap<>();
 
-    @SuppressWarnings("deprecation") // TODO: remove in follow up PR when `RETRIES` is removed
-    public InternalTopicManager(final Admin adminClient, final StreamsConfig streamsConfig) {
+    public InternalTopicManager(final Time time,
+                                final Admin adminClient,
+                                final StreamsConfig streamsConfig) {
+        this.time = time;
         this.adminClient = adminClient;
 
         final LogContext logContext = new LogContext(String.format("stream-thread [%s] ", Thread.currentThread().getName()));
         log = logContext.logger(getClass());
 
         replicationFactor = streamsConfig.getInt(StreamsConfig.REPLICATION_FACTOR_CONFIG).shortValue();
         windowChangeLogAdditionalRetention = streamsConfig.getLong(StreamsConfig.WINDOW_STORE_CHANGE_LOG_ADDITIONAL_RETENTION_MS_CONFIG);
-        final AdminClientConfig adminConfigs = new ClientUtils.QuietAdminClientConfig(streamsConfig);
-        retries = adminConfigs.getInt(AdminClientConfig.RETRIES_CONFIG);
-        retryBackOffMs = adminConfigs.getLong(AdminClientConfig.RETRY_BACKOFF_MS_CONFIG);
+        retryBackOffMs = streamsConfig.getLong(StreamsConfig.RETRY_BACKOFF_MS_CONFIG);
+        final Map<String, Object> consumerConfig = streamsConfig.getMainConsumerConfigs("dummy", "dummy", -1);
+        // need to add mandatory configs; otherwise `QuietConsumerConfig` throws
+        consumerConfig.put(ConsumerConfig.KEY_DESERIALIZER_CLASS_CONFIG, ByteArrayDeserializer.class);
+        consumerConfig.put(ConsumerConfig.VALUE_DESERIALIZER_CLASS_CONFIG, ByteArrayDeserializer.class);
+        retryTimeoutMs = new QuietConsumerConfig(consumerConfig).getInt(ConsumerConfig.MAX_POLL_INTERVAL_MS_CONFIG) / 2L;

Review comment:
       Today since we do not have ways to partially create tasks we'd have to create all topics to make sure all tasks are "complete" within each rebalance, if we cannot successfully create the topics within the poll.interval (i.e. we'd need to complete that rebalance with the poll.interval, and I guess halving it is to be more conservative), then killing that thread is not very useful anyways since we cannot proceed with the initializable tasks anyways.
   
   That being said, with the upcoming work I'd agree that just shutdown the thread and allow users to optionally retry rebalance with new threads would be preferrable.




----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org