ashmeet13 commented on code in PR #12988:
URL: https://github.com/apache/kafka/pull/12988#discussion_r1721230587


##########
streams/src/main/java/org/apache/kafka/streams/StreamsConfig.java:
##########
@@ -1182,41 +1171,59 @@ public class StreamsConfig extends AbstractConfig {
                     WINDOW_SIZE_MS_DOC);
     }
 
-    // this is the list of configs for underlying clients
-    // that streams prefer different default values
-    private static final Map<String, Object> PRODUCER_DEFAULT_OVERRIDES;
+    private static final Map<String, Object> KS_DEFAULT_PRODUCER_CONFIGS;
     static {
         final Map<String, Object> tempProducerDefaultOverrides = new 
HashMap<>();
         tempProducerDefaultOverrides.put(ProducerConfig.LINGER_MS_CONFIG, 
"100");
-        PRODUCER_DEFAULT_OVERRIDES = 
Collections.unmodifiableMap(tempProducerDefaultOverrides);
+
+        KS_DEFAULT_PRODUCER_CONFIGS = 
Collections.unmodifiableMap(tempProducerDefaultOverrides);
     }
 
-    private static final Map<String, Object> PRODUCER_EOS_OVERRIDES;
+    private static final Map<String, Object> 
KS_DEFAULT_PRODUCER_CONFIGS_EOS_ENABLED;
     static {
-        final Map<String, Object> tempProducerDefaultOverrides = new 
HashMap<>(PRODUCER_DEFAULT_OVERRIDES);
+        final Map<String, Object> tempProducerDefaultOverrides = new 
HashMap<>(KS_DEFAULT_PRODUCER_CONFIGS);
         
tempProducerDefaultOverrides.put(ProducerConfig.DELIVERY_TIMEOUT_MS_CONFIG, 
Integer.MAX_VALUE);
-        
tempProducerDefaultOverrides.put(ProducerConfig.ENABLE_IDEMPOTENCE_CONFIG, 
true);
         // Reduce the transaction timeout for quicker pending offset 
expiration on broker side.
         
tempProducerDefaultOverrides.put(ProducerConfig.TRANSACTION_TIMEOUT_CONFIG, 
DEFAULT_TRANSACTION_TIMEOUT);
 
-        PRODUCER_EOS_OVERRIDES = 
Collections.unmodifiableMap(tempProducerDefaultOverrides);
+        KS_DEFAULT_PRODUCER_CONFIGS_EOS_ENABLED = 
Collections.unmodifiableMap(tempProducerDefaultOverrides);
+    }
+
+    private static final Map<String, Object> 
KS_CONTROLLED_PRODUCER_CONFIGS_EOS_ENABLED;
+    static {
+        final Map<String, Object> tempProducerDefaultOverrides = new 
HashMap<>();
+        
tempProducerDefaultOverrides.put(ProducerConfig.ENABLE_IDEMPOTENCE_CONFIG, 
true);
+        
tempProducerDefaultOverrides.put(ProducerConfig.TRANSACTIONAL_ID_CONFIG, null);
+
+        KS_CONTROLLED_PRODUCER_CONFIGS_EOS_ENABLED = 
Collections.unmodifiableMap(tempProducerDefaultOverrides);
     }
 
-    private static final Map<String, Object> CONSUMER_DEFAULT_OVERRIDES;
+    private static final Map<String, Object> KS_DEFAULT_CONSUMER_CONFIGS;
     static {
         final Map<String, Object> tempConsumerDefaultOverrides = new 
HashMap<>();
         
tempConsumerDefaultOverrides.put(ConsumerConfig.MAX_POLL_RECORDS_CONFIG, 
"1000");
         
tempConsumerDefaultOverrides.put(ConsumerConfig.AUTO_OFFSET_RESET_CONFIG, 
"earliest");
-        
tempConsumerDefaultOverrides.put(ConsumerConfig.ENABLE_AUTO_COMMIT_CONFIG, 
"false");
         tempConsumerDefaultOverrides.put("internal.leave.group.on.close", 
false);
-        CONSUMER_DEFAULT_OVERRIDES = 
Collections.unmodifiableMap(tempConsumerDefaultOverrides);
+
+        KS_DEFAULT_CONSUMER_CONFIGS = 
Collections.unmodifiableMap(tempConsumerDefaultOverrides);
+    }
+
+    private static final Map<String, Object> KS_CONTROLLED_CONSUMER_CONFIGS;

Review Comment:
   Working on this change



##########
streams/src/main/java/org/apache/kafka/streams/StreamsConfig.java:
##########
@@ -1609,6 +1605,8 @@ public Map<String, Object> getMainConsumerConfigs(final 
String groupId, final St
         final Map<String, Object> mainConsumerProps = 
originalsWithPrefix(MAIN_CONSUMER_PREFIX);
         consumerProps.putAll(mainConsumerProps);
 
+        validateConsumerPropertyMap(consumerProps);

Review Comment:
   Working on this change



##########
streams/src/main/java/org/apache/kafka/streams/StreamsConfig.java:
##########
@@ -1714,6 +1714,8 @@ public Map<String, Object> getGlobalConsumerConfigs(final 
String clientId) {
         final Map<String, Object> globalConsumerProps = 
originalsWithPrefix(GLOBAL_CONSUMER_PREFIX);
         baseConsumerProps.putAll(globalConsumerProps);
 
+        validateConsumerPropertyMap(baseConsumerProps);
+
         // no need to set group id for a global consumer
         baseConsumerProps.remove(ConsumerConfig.GROUP_ID_CONFIG);

Review Comment:
   Working on this change



-- 
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.

To unsubscribe, e-mail: [email protected]

For queries about this service, please contact Infrastructure at:
[email protected]

Reply via email to