[ https://issues.apache.org/jira/browse/BEAM-6285?focusedWorklogId=189076&page=com.atlassian.jira.plugin.system.issuetabpanels:worklog-tabpanel#worklog-189076 ]
ASF GitHub Bot logged work on BEAM-6285: ---------------------------------------- Author: ASF GitHub Bot Created on: 23/Jan/19 17:22 Start Date: 23/Jan/19 17:22 Worklog Time Spent: 10m Work Description: rangadi commented on pull request #7598: [BEAM-6285] add parameters for offsetConsumer in KafkaIO.read() URL: https://github.com/apache/beam/pull/7598#discussion_r250292351 ########## File path: sdks/java/io/kafka/src/main/java/org/apache/beam/sdk/io/kafka/KafkaIO.java ########## @@ -656,6 +661,28 @@ return toBuilder().setCommitOffsetsInFinalizeEnabled(true).build(); } + /** + * Set additional parameters for KafkaUnboundedReader.offsetConsumer, added by BEAM-6285. + * + * <p>In KafkaIO.read(), there're two consumers running in the backend actually:<br> + * 1. KafkaUnboundedReader.consumer, which is the main consumer to read data from kafka + * topic(s);<br> + * 2. KafkaUnboundedReader.offsetConsumer, which is used to estimate backlog, by fetching the + * latest offset;<br> + * + * <p>By default, KafkaUnboundedReader.offsetConsumer shares the same parameters as + * KafkaUnboundedReader.consumer, with a generated ConsumerConfig.GROUP_ID_CONFIG(value is + * "%topicname_offset_consumer_%randonNumber_%groupId"). This may not work when the topic is + * secured in Kafka. + * + * <p>In this case with secured topics, you can add specific parameters for + * KafkaUnboundedReader.offsetConsumer, to overwrite parameters from + * KafkaUnboundedReader.consumer. + */ + public Read<K, V> withOffsetConsumerConfig(Map<String, Object> offsetConsumerConfig) { Review comment: Rename this to `withOffsetConsumerConfigOverrides()`, that way it is more explicit to the user that it first derives its config from the main consumer. ---------------------------------------------------------------- This is an automated message from the Apache Git Service. To respond to the message, please log on GitHub and use the URL above to go to the specific comment. For queries about this service, please contact Infrastructure at: us...@infra.apache.org Issue Time Tracking ------------------- Worklog Id: (was: 189076) Time Spent: 50m (was: 40m) > add parameters for offsetConsumer in KafkaIO.read() > --------------------------------------------------- > > Key: BEAM-6285 > URL: https://issues.apache.org/jira/browse/BEAM-6285 > Project: Beam > Issue Type: Improvement > Components: io-java-kafka > Reporter: Xu Mingmin > Assignee: Xu Mingmin > Priority: Major > Time Spent: 50m > Remaining Estimate: 0h > > Add an option in KafkaIO.read() to avoid using `offsetConsumer`. > *What's the problem:* > When security is enabled, function > `KafkaUnboundedReader.updateLatestOffsets()` always fails with lots of WARN > log. > *What's the cause:* > `offsetConsumer` uses a mocked `ConsumerConfig.GROUP_ID_CONFIG`, and it > doesn't work as expected when security is enabled. In our case, > `ConsumerConfig.GROUP_ID_CONFIG` is the key to validate whether this consumer > is approved. > *What's the solution:* > Add an option to disable `offsetConsumer` by developers; > *What's the side effect:* > With `offsetConsumer` disabled, backlog is unknown. > > [~rangadi] any comments? > > -- This message was sent by Atlassian JIRA (v7.6.3#76005)