ashmeet13 commented on code in PR #12988:
URL: https://github.com/apache/kafka/pull/12988#discussion_r1720931028
##########
streams/src/main/java/org/apache/kafka/streams/StreamsConfig.java:
##########
@@ -1182,41 +1175,64 @@ 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;
+ // KS_DEFAULT_PRODUCER_CONFIGS - default producer configs for Kafka Streams
+ 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;
+ // KS_DEFAULT_PRODUCER_CONFIGS_EOS_ENABLED - default producer configs for
Kafka Streams with EOS enabled
+ 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);
+ }
+
+ // KS_CONTROLLED_PRODUCER_CONFIGS_EOS_ENABLED - Kafka Streams producer
configs that cannot be overridden by the user with EOS enabled
+ 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;
+ // KS_DEFAULT_CONSUMER_CONFIGS - default consumer configs for Kafka Streams
+ 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> CONSUMER_EOS_OVERRIDES;
+ // KS_CONTROLLED_CONSUMER_CONFIGS - Kafka Streams consumer configs that
cannot be overridden by the user
Review Comment:
Removed. Resolving.
--
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]