Re: Review Request 19731: Patch for KAFKA-1328
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/19731/#review44964 --- If a consumer fails to establish a connection to zookeeper, how is that indicated? Also, just to confirm I understand the unsubscribe API correctly: if I have a large number of ephemeral consumers, will unsubscribe remove the consumer from zookeeper? Is that what the TODO: rebalance means? - Baran Nohutcuoglu On May 20, 2014, 11:34 p.m., Neha Narkhede wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/19731/ --- (Updated May 20, 2014, 11:34 p.m.) Review request for kafka. Bugs: KAFKA-1328 https://issues.apache.org/jira/browse/KAFKA-1328 Repository: kafka Description --- Fixed inconsistent javadoc for position(), committed() and offsetsBeforeTime() Converted ellipsis to Collection in a bunch of places, removed TimeUnit from the poll() API 1. Improved documentation on the position() API 2. Changed signature of commit API to remove Future and include a sync flag Included Jun's review suggestions part 2, except change to the commit() API since it needs more thought Review comments from Jun and Guozhang Checked in ConsumerRecordMetadata Fixed the javadoc usage examples in KafkaConsumer to match the API changes Changed the signature of poll to return MapString,ConsumerRecordMetadata to organize the ConsumerRecords around topic and then optionally around partition. This will serve the group management as well as custom partition subscription use cases 1. Changed the signature of poll() to return MapString, ListConsumerRecord 2. Changed ConsumerRecord to throw an exception if an error is detected for the partition. For example, if a single large message is larger than the total memory just for that partition, we don't want poll() to throw an exception since that will affect the processing of the remaining partitions as well Fixed MockConsumer to make subscribe(topics) and subscribe(partitions) mutually exclusive Changed the package to org.apache.kafka.clients.consumer from kafka.clients.consumer Changed the package to org.apache.kafka.clients.consumer from kafka.clients.consumer 1. Removed the commitAsync() APIs 2. Changed the commit() APIs to return a Future Fixed configs to match the producer side configs for metrics Renamed AUTO_COMMIT_ENABLE_CONFIG to ENABLE_AUTO_COMMIT_CONFIG Addressing review comments from Tim and Guozhang Rebasing after producer side config cleanup Added license headers Cleaned javadoc for ConsumerConfig Fixed minor indentation in ConsumerConfig Improve docs on ConsumerConfig 1. Added ClientUtils 2. Added basic constructor implementation for KafkaConsumer Improved MockConsumer Chris's feedback and also consumer rewind example code Added commit() and commitAsync() APIs to the consumer and updated docs and examples to reflect that 1. Added consumer usage examples to javadoc 2. Changed signature of APIs that accept or return offsets from list of offsets to map of offsets Improved example for using ConsumerRebalanceCallback Improved example for using ConsumerRebalanceCallback Included Jun's review comments and renamed positions to seek. Also included position() Changes to javadoc for positions() Changed the javadoc for ConsumerRebalanceCallback Changing unsubscribe to also take in var args for topic list Incorporated first round of feedback from Jay, Pradeep and Mattijs on the mailing list Updated configs Javadoc for consumer complete Completed docs for Consumer and ConsumerRebalanceCallback. Added MockConsumer Added the initial interfaces and related documentation for the consumer. More docs required to complete the public API Diffs - clients/src/main/java/org/apache/kafka/clients/consumer/Consumer.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerConfig.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRebalanceCallback.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRecord.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRecords.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/KafkaConsumer.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/MockConsumer.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/OffsetMetadata.java PRE-CREATION
Re: Review Request 19731: Patch for KAFKA-1328
On May 19, 2014, 5:30 p.m., Jun Rao wrote: clients/src/main/java/org/apache/kafka/clients/consumer/Consumer.java, lines 70-71 https://reviews.apache.org/r/19731/diff/11/?file=583622#file583622line70 Instead of introducing ConsumerRecords, I'd prefer just returning MapString, ListConsumerRecord. Guozhang Wang wrote: The question I think we are trying to answer is how to expose the per-partition error code back to the user. So far it seems we do not have an ideal solution yet. Agree with Guozhang. For that, it's worth keeping it the way it is. On May 19, 2014, 5:30 p.m., Jun Rao wrote: clients/src/main/java/org/apache/kafka/clients/consumer/Consumer.java, lines 99-119 https://reviews.apache.org/r/19731/diff/11/?file=583622#file583622line99 Since these apis have Map in the return value, they are really intended as a batch api. So, would it better to have the input as a set of TopicPartitions? This will also make sure that the passed in partitions are unique. We discussed this offline, changing it to Collection. On May 19, 2014, 5:30 p.m., Jun Rao wrote: clients/src/main/java/org/apache/kafka/clients/consumer/internals/FutureOffsetMetadata.java, lines 1-36 https://reviews.apache.org/r/19731/diff/11/?file=583630#file583630line1 Do we still need this class? Removed. On May 19, 2014, 5:30 p.m., Jun Rao wrote: clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRebalanceCallback.java, lines 36-47 https://reviews.apache.org/r/19731/diff/11/?file=583624#file583624line36 Would it be better to use Set[TopicPartition] instead of ellipsis? This will make it clear that they are unique. Changing to Collection. On May 19, 2014, 5:30 p.m., Jun Rao wrote: clients/src/main/java/org/apache/kafka/clients/consumer/Consumer.java, lines 84-87 https://reviews.apache.org/r/19731/diff/11/?file=583622#file583622line84 It's going to be a bit confusing to the caller to expect both an error code in the return value and an exception. It seems that we can just translate exceptions into error codes. In async mode, the return value will be null. So it's impossible for the caller to get the error code. However, by choosing the async mode, the caller doesn't assume the commit to succeed immediately. It's probably ok just to try to commit again when it's called. Makes sense. Changed the javadoc to reflect that it does not throw any exception - Neha --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/19731/#review43365 --- On May 16, 2014, 6:46 p.m., Neha Narkhede wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/19731/ --- (Updated May 16, 2014, 6:46 p.m.) Review request for kafka. Bugs: KAFKA-1328 https://issues.apache.org/jira/browse/KAFKA-1328 Repository: kafka Description --- 1. Improved documentation on the position() API 2. Changed signature of commit API to remove Future and include a sync flag Included Jun's review suggestions part 2, except change to the commit() API since it needs more thought Review comments from Jun and Guozhang Checked in ConsumerRecordMetadata Fixed the javadoc usage examples in KafkaConsumer to match the API changes Changed the signature of poll to return MapString,ConsumerRecordMetadata to organize the ConsumerRecords around topic and then optionally around partition. This will serve the group management as well as custom partition subscription use cases 1. Changed the signature of poll() to return MapString, ListConsumerRecord 2. Changed ConsumerRecord to throw an exception if an error is detected for the partition. For example, if a single large message is larger than the total memory just for that partition, we don't want poll() to throw an exception since that will affect the processing of the remaining partitions as well Fixed MockConsumer to make subscribe(topics) and subscribe(partitions) mutually exclusive Changed the package to org.apache.kafka.clients.consumer from kafka.clients.consumer Changed the package to org.apache.kafka.clients.consumer from kafka.clients.consumer 1. Removed the commitAsync() APIs 2. Changed the commit() APIs to return a Future Fixed configs to match the producer side configs for metrics Renamed AUTO_COMMIT_ENABLE_CONFIG to ENABLE_AUTO_COMMIT_CONFIG Addressing review comments from Tim and Guozhang Rebasing after producer side config cleanup Added license headers Cleaned javadoc for ConsumerConfig Fixed minor indentation in ConsumerConfig Improve docs on
Re: Review Request 19731: Patch for KAFKA-1328
On May 19, 2014, 6:22 p.m., Guozhang Wang wrote: clients/src/main/java/org/apache/kafka/clients/consumer/Consumer.java, lines 99-119 https://reviews.apache.org/r/19731/diff/11/?file=583622#file583622line99 Should we discuss about code conventions of using ellipse (i.e. an array list) versus using Set, since Jun's reason here also applies to subscribe/unsubscribe? It depends on the nature of usage. It is inconvenient to use subscribe/unsubscribe with a Set (requires multiple lines of code). Also this call modifies in memory state so it doesn't need to be batched. Since performance is not a concern and it aids usability, I would argue to keep the ellipses in subscribe/unsubscribe and probably change it to Collection in other places. On May 19, 2014, 6:22 p.m., Guozhang Wang wrote: clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRecords.java, lines 39-53 https://reviews.apache.org/r/19731/diff/11/?file=583626#file583626line39 I thought originally this class is used to wrap and expose the per-partition error code so that we do not need to throw an exception on every record of that partition. Is that still true? The purpose of this class is to return the records grouped easily by topic as well as (optionally) a partition - Neha --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/19731/#review43385 --- On May 16, 2014, 6:46 p.m., Neha Narkhede wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/19731/ --- (Updated May 16, 2014, 6:46 p.m.) Review request for kafka. Bugs: KAFKA-1328 https://issues.apache.org/jira/browse/KAFKA-1328 Repository: kafka Description --- 1. Improved documentation on the position() API 2. Changed signature of commit API to remove Future and include a sync flag Included Jun's review suggestions part 2, except change to the commit() API since it needs more thought Review comments from Jun and Guozhang Checked in ConsumerRecordMetadata Fixed the javadoc usage examples in KafkaConsumer to match the API changes Changed the signature of poll to return MapString,ConsumerRecordMetadata to organize the ConsumerRecords around topic and then optionally around partition. This will serve the group management as well as custom partition subscription use cases 1. Changed the signature of poll() to return MapString, ListConsumerRecord 2. Changed ConsumerRecord to throw an exception if an error is detected for the partition. For example, if a single large message is larger than the total memory just for that partition, we don't want poll() to throw an exception since that will affect the processing of the remaining partitions as well Fixed MockConsumer to make subscribe(topics) and subscribe(partitions) mutually exclusive Changed the package to org.apache.kafka.clients.consumer from kafka.clients.consumer Changed the package to org.apache.kafka.clients.consumer from kafka.clients.consumer 1. Removed the commitAsync() APIs 2. Changed the commit() APIs to return a Future Fixed configs to match the producer side configs for metrics Renamed AUTO_COMMIT_ENABLE_CONFIG to ENABLE_AUTO_COMMIT_CONFIG Addressing review comments from Tim and Guozhang Rebasing after producer side config cleanup Added license headers Cleaned javadoc for ConsumerConfig Fixed minor indentation in ConsumerConfig Improve docs on ConsumerConfig 1. Added ClientUtils 2. Added basic constructor implementation for KafkaConsumer Improved MockConsumer Chris's feedback and also consumer rewind example code Added commit() and commitAsync() APIs to the consumer and updated docs and examples to reflect that 1. Added consumer usage examples to javadoc 2. Changed signature of APIs that accept or return offsets from list of offsets to map of offsets Improved example for using ConsumerRebalanceCallback Improved example for using ConsumerRebalanceCallback Included Jun's review comments and renamed positions to seek. Also included position() Changes to javadoc for positions() Changed the javadoc for ConsumerRebalanceCallback Changing unsubscribe to also take in var args for topic list Incorporated first round of feedback from Jay, Pradeep and Mattijs on the mailing list Updated configs Javadoc for consumer complete Completed docs for Consumer and ConsumerRebalanceCallback. Added MockConsumer Added the initial interfaces and related documentation for the consumer. More docs required to complete the public API Diffs
Re: Review Request 19731: Patch for KAFKA-1328
On May 19, 2014, 6:51 p.m., Jun Rao wrote: clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerConfig.java, lines 27-28 https://reviews.apache.org/r/19731/diff/11/?file=583623#file583623line27 Have we added the config for selecting the strategy for assigning partitions? yes - Neha --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/19731/#review43395 --- On May 20, 2014, 10:55 p.m., Neha Narkhede wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/19731/ --- (Updated May 20, 2014, 10:55 p.m.) Review request for kafka. Bugs: KAFKA-1328 https://issues.apache.org/jira/browse/KAFKA-1328 Repository: kafka Description --- Converted ellipsis to Collection in a bunch of places, removed TimeUnit from the poll() API 1. Improved documentation on the position() API 2. Changed signature of commit API to remove Future and include a sync flag Included Jun's review suggestions part 2, except change to the commit() API since it needs more thought Review comments from Jun and Guozhang Checked in ConsumerRecordMetadata Fixed the javadoc usage examples in KafkaConsumer to match the API changes Changed the signature of poll to return MapString,ConsumerRecordMetadata to organize the ConsumerRecords around topic and then optionally around partition. This will serve the group management as well as custom partition subscription use cases 1. Changed the signature of poll() to return MapString, ListConsumerRecord 2. Changed ConsumerRecord to throw an exception if an error is detected for the partition. For example, if a single large message is larger than the total memory just for that partition, we don't want poll() to throw an exception since that will affect the processing of the remaining partitions as well Fixed MockConsumer to make subscribe(topics) and subscribe(partitions) mutually exclusive Changed the package to org.apache.kafka.clients.consumer from kafka.clients.consumer Changed the package to org.apache.kafka.clients.consumer from kafka.clients.consumer 1. Removed the commitAsync() APIs 2. Changed the commit() APIs to return a Future Fixed configs to match the producer side configs for metrics Renamed AUTO_COMMIT_ENABLE_CONFIG to ENABLE_AUTO_COMMIT_CONFIG Addressing review comments from Tim and Guozhang Rebasing after producer side config cleanup Added license headers Cleaned javadoc for ConsumerConfig Fixed minor indentation in ConsumerConfig Improve docs on ConsumerConfig 1. Added ClientUtils 2. Added basic constructor implementation for KafkaConsumer Improved MockConsumer Chris's feedback and also consumer rewind example code Added commit() and commitAsync() APIs to the consumer and updated docs and examples to reflect that 1. Added consumer usage examples to javadoc 2. Changed signature of APIs that accept or return offsets from list of offsets to map of offsets Improved example for using ConsumerRebalanceCallback Improved example for using ConsumerRebalanceCallback Included Jun's review comments and renamed positions to seek. Also included position() Changes to javadoc for positions() Changed the javadoc for ConsumerRebalanceCallback Changing unsubscribe to also take in var args for topic list Incorporated first round of feedback from Jay, Pradeep and Mattijs on the mailing list Updated configs Javadoc for consumer complete Completed docs for Consumer and ConsumerRebalanceCallback. Added MockConsumer Added the initial interfaces and related documentation for the consumer. More docs required to complete the public API Diffs - clients/src/main/java/org/apache/kafka/clients/consumer/Consumer.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerConfig.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRebalanceCallback.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRecord.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRecords.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/KafkaConsumer.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/MockConsumer.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/OffsetMetadata.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/producer/KafkaProducer.java 90cacbd8941b7c8f15d1417c821945c1ac1b4d00
Re: Review Request 19731: Patch for KAFKA-1328
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/19731/ --- (Updated May 20, 2014, 11:34 p.m.) Review request for kafka. Bugs: KAFKA-1328 https://issues.apache.org/jira/browse/KAFKA-1328 Repository: kafka Description (updated) --- Fixed inconsistent javadoc for position(), committed() and offsetsBeforeTime() Converted ellipsis to Collection in a bunch of places, removed TimeUnit from the poll() API 1. Improved documentation on the position() API 2. Changed signature of commit API to remove Future and include a sync flag Included Jun's review suggestions part 2, except change to the commit() API since it needs more thought Review comments from Jun and Guozhang Checked in ConsumerRecordMetadata Fixed the javadoc usage examples in KafkaConsumer to match the API changes Changed the signature of poll to return MapString,ConsumerRecordMetadata to organize the ConsumerRecords around topic and then optionally around partition. This will serve the group management as well as custom partition subscription use cases 1. Changed the signature of poll() to return MapString, ListConsumerRecord 2. Changed ConsumerRecord to throw an exception if an error is detected for the partition. For example, if a single large message is larger than the total memory just for that partition, we don't want poll() to throw an exception since that will affect the processing of the remaining partitions as well Fixed MockConsumer to make subscribe(topics) and subscribe(partitions) mutually exclusive Changed the package to org.apache.kafka.clients.consumer from kafka.clients.consumer Changed the package to org.apache.kafka.clients.consumer from kafka.clients.consumer 1. Removed the commitAsync() APIs 2. Changed the commit() APIs to return a Future Fixed configs to match the producer side configs for metrics Renamed AUTO_COMMIT_ENABLE_CONFIG to ENABLE_AUTO_COMMIT_CONFIG Addressing review comments from Tim and Guozhang Rebasing after producer side config cleanup Added license headers Cleaned javadoc for ConsumerConfig Fixed minor indentation in ConsumerConfig Improve docs on ConsumerConfig 1. Added ClientUtils 2. Added basic constructor implementation for KafkaConsumer Improved MockConsumer Chris's feedback and also consumer rewind example code Added commit() and commitAsync() APIs to the consumer and updated docs and examples to reflect that 1. Added consumer usage examples to javadoc 2. Changed signature of APIs that accept or return offsets from list of offsets to map of offsets Improved example for using ConsumerRebalanceCallback Improved example for using ConsumerRebalanceCallback Included Jun's review comments and renamed positions to seek. Also included position() Changes to javadoc for positions() Changed the javadoc for ConsumerRebalanceCallback Changing unsubscribe to also take in var args for topic list Incorporated first round of feedback from Jay, Pradeep and Mattijs on the mailing list Updated configs Javadoc for consumer complete Completed docs for Consumer and ConsumerRebalanceCallback. Added MockConsumer Added the initial interfaces and related documentation for the consumer. More docs required to complete the public API Diffs (updated) - clients/src/main/java/org/apache/kafka/clients/consumer/Consumer.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerConfig.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRebalanceCallback.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRecord.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRecords.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/KafkaConsumer.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/MockConsumer.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/OffsetMetadata.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/producer/KafkaProducer.java 90cacbd8941b7c8f15d1417c821945c1ac1b4d00 clients/src/main/java/org/apache/kafka/common/utils/ClientUtils.java PRE-CREATION clients/src/test/java/org/apache/kafka/clients/consumer/ConsumerExampleTest.java PRE-CREATION Diff: https://reviews.apache.org/r/19731/diff/ Testing --- Thanks, Neha Narkhede
Re: Review Request 19731: Patch for KAFKA-1328
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/19731/#review43558 --- Ship it! clients/src/main/java/org/apache/kafka/clients/consumer/Consumer.java https://reviews.apache.org/r/19731/#comment77768 Could we add in the comment that the timeout is in ms? - Jun Rao On May 20, 2014, 11:34 p.m., Neha Narkhede wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/19731/ --- (Updated May 20, 2014, 11:34 p.m.) Review request for kafka. Bugs: KAFKA-1328 https://issues.apache.org/jira/browse/KAFKA-1328 Repository: kafka Description --- Fixed inconsistent javadoc for position(), committed() and offsetsBeforeTime() Converted ellipsis to Collection in a bunch of places, removed TimeUnit from the poll() API 1. Improved documentation on the position() API 2. Changed signature of commit API to remove Future and include a sync flag Included Jun's review suggestions part 2, except change to the commit() API since it needs more thought Review comments from Jun and Guozhang Checked in ConsumerRecordMetadata Fixed the javadoc usage examples in KafkaConsumer to match the API changes Changed the signature of poll to return MapString,ConsumerRecordMetadata to organize the ConsumerRecords around topic and then optionally around partition. This will serve the group management as well as custom partition subscription use cases 1. Changed the signature of poll() to return MapString, ListConsumerRecord 2. Changed ConsumerRecord to throw an exception if an error is detected for the partition. For example, if a single large message is larger than the total memory just for that partition, we don't want poll() to throw an exception since that will affect the processing of the remaining partitions as well Fixed MockConsumer to make subscribe(topics) and subscribe(partitions) mutually exclusive Changed the package to org.apache.kafka.clients.consumer from kafka.clients.consumer Changed the package to org.apache.kafka.clients.consumer from kafka.clients.consumer 1. Removed the commitAsync() APIs 2. Changed the commit() APIs to return a Future Fixed configs to match the producer side configs for metrics Renamed AUTO_COMMIT_ENABLE_CONFIG to ENABLE_AUTO_COMMIT_CONFIG Addressing review comments from Tim and Guozhang Rebasing after producer side config cleanup Added license headers Cleaned javadoc for ConsumerConfig Fixed minor indentation in ConsumerConfig Improve docs on ConsumerConfig 1. Added ClientUtils 2. Added basic constructor implementation for KafkaConsumer Improved MockConsumer Chris's feedback and also consumer rewind example code Added commit() and commitAsync() APIs to the consumer and updated docs and examples to reflect that 1. Added consumer usage examples to javadoc 2. Changed signature of APIs that accept or return offsets from list of offsets to map of offsets Improved example for using ConsumerRebalanceCallback Improved example for using ConsumerRebalanceCallback Included Jun's review comments and renamed positions to seek. Also included position() Changes to javadoc for positions() Changed the javadoc for ConsumerRebalanceCallback Changing unsubscribe to also take in var args for topic list Incorporated first round of feedback from Jay, Pradeep and Mattijs on the mailing list Updated configs Javadoc for consumer complete Completed docs for Consumer and ConsumerRebalanceCallback. Added MockConsumer Added the initial interfaces and related documentation for the consumer. More docs required to complete the public API Diffs - clients/src/main/java/org/apache/kafka/clients/consumer/Consumer.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerConfig.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRebalanceCallback.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRecord.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRecords.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/KafkaConsumer.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/MockConsumer.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/OffsetMetadata.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/producer/KafkaProducer.java 90cacbd8941b7c8f15d1417c821945c1ac1b4d00
Re: Review Request 19731: Patch for KAFKA-1328
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/19731/#review43365 --- clients/src/main/java/org/apache/kafka/clients/consumer/Consumer.java https://reviews.apache.org/r/19731/#comment77450 This api has to be implemented on top of java Selector, which only supports milli-secs time unit when calling select(). So, we probably should just assume that the timeout is in milli-secs and remove timeUnit. clients/src/main/java/org/apache/kafka/clients/consumer/Consumer.java https://reviews.apache.org/r/19731/#comment77454 Instead of introducing ConsumerRecords, I'd prefer just returning MapString, ListConsumerRecord. clients/src/main/java/org/apache/kafka/clients/consumer/Consumer.java https://reviews.apache.org/r/19731/#comment77451 It's going to be a bit confusing to the caller to expect both an error code in the return value and an exception. It seems that we can just translate exceptions into error codes. In async mode, the return value will be null. So it's impossible for the caller to get the error code. However, by choosing the async mode, the caller doesn't assume the commit to succeed immediately. It's probably ok just to try to commit again when it's called. clients/src/main/java/org/apache/kafka/clients/consumer/Consumer.java https://reviews.apache.org/r/19731/#comment77452 Since these apis have Map in the return value, they are really intended as a batch api. So, would it better to have the input as a set of TopicPartitions? This will also make sure that the passed in partitions are unique. clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRebalanceCallback.java https://reviews.apache.org/r/19731/#comment77453 Would it be better to use Set[TopicPartition] instead of ellipsis? This will make it clear that they are unique. clients/src/main/java/org/apache/kafka/clients/consumer/OffsetMetadata.java https://reviews.apache.org/r/19731/#comment77455 To be consistent with ConsumerRecord, we probably should just return Exception. clients/src/main/java/org/apache/kafka/clients/consumer/internals/FutureOffsetMetadata.java https://reviews.apache.org/r/19731/#comment77456 Do we still need this class? - Jun Rao On May 16, 2014, 6:46 p.m., Neha Narkhede wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/19731/ --- (Updated May 16, 2014, 6:46 p.m.) Review request for kafka. Bugs: KAFKA-1328 https://issues.apache.org/jira/browse/KAFKA-1328 Repository: kafka Description --- 1. Improved documentation on the position() API 2. Changed signature of commit API to remove Future and include a sync flag Included Jun's review suggestions part 2, except change to the commit() API since it needs more thought Review comments from Jun and Guozhang Checked in ConsumerRecordMetadata Fixed the javadoc usage examples in KafkaConsumer to match the API changes Changed the signature of poll to return MapString,ConsumerRecordMetadata to organize the ConsumerRecords around topic and then optionally around partition. This will serve the group management as well as custom partition subscription use cases 1. Changed the signature of poll() to return MapString, ListConsumerRecord 2. Changed ConsumerRecord to throw an exception if an error is detected for the partition. For example, if a single large message is larger than the total memory just for that partition, we don't want poll() to throw an exception since that will affect the processing of the remaining partitions as well Fixed MockConsumer to make subscribe(topics) and subscribe(partitions) mutually exclusive Changed the package to org.apache.kafka.clients.consumer from kafka.clients.consumer Changed the package to org.apache.kafka.clients.consumer from kafka.clients.consumer 1. Removed the commitAsync() APIs 2. Changed the commit() APIs to return a Future Fixed configs to match the producer side configs for metrics Renamed AUTO_COMMIT_ENABLE_CONFIG to ENABLE_AUTO_COMMIT_CONFIG Addressing review comments from Tim and Guozhang Rebasing after producer side config cleanup Added license headers Cleaned javadoc for ConsumerConfig Fixed minor indentation in ConsumerConfig Improve docs on ConsumerConfig 1. Added ClientUtils 2. Added basic constructor implementation for KafkaConsumer Improved MockConsumer Chris's feedback and also consumer rewind example code Added commit() and commitAsync() APIs to the consumer and updated docs and examples to reflect that 1. Added consumer usage examples to javadoc 2. Changed
Re: Review Request 19731: Patch for KAFKA-1328
On May 19, 2014, 5:30 p.m., Jun Rao wrote: clients/src/main/java/org/apache/kafka/clients/consumer/Consumer.java, lines 70-71 https://reviews.apache.org/r/19731/diff/11/?file=583622#file583622line70 Instead of introducing ConsumerRecords, I'd prefer just returning MapString, ListConsumerRecord. The question I think we are trying to answer is how to expose the per-partition error code back to the user. So far it seems we do not have an ideal solution yet. - Guozhang --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/19731/#review43365 --- On May 16, 2014, 6:46 p.m., Neha Narkhede wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/19731/ --- (Updated May 16, 2014, 6:46 p.m.) Review request for kafka. Bugs: KAFKA-1328 https://issues.apache.org/jira/browse/KAFKA-1328 Repository: kafka Description --- 1. Improved documentation on the position() API 2. Changed signature of commit API to remove Future and include a sync flag Included Jun's review suggestions part 2, except change to the commit() API since it needs more thought Review comments from Jun and Guozhang Checked in ConsumerRecordMetadata Fixed the javadoc usage examples in KafkaConsumer to match the API changes Changed the signature of poll to return MapString,ConsumerRecordMetadata to organize the ConsumerRecords around topic and then optionally around partition. This will serve the group management as well as custom partition subscription use cases 1. Changed the signature of poll() to return MapString, ListConsumerRecord 2. Changed ConsumerRecord to throw an exception if an error is detected for the partition. For example, if a single large message is larger than the total memory just for that partition, we don't want poll() to throw an exception since that will affect the processing of the remaining partitions as well Fixed MockConsumer to make subscribe(topics) and subscribe(partitions) mutually exclusive Changed the package to org.apache.kafka.clients.consumer from kafka.clients.consumer Changed the package to org.apache.kafka.clients.consumer from kafka.clients.consumer 1. Removed the commitAsync() APIs 2. Changed the commit() APIs to return a Future Fixed configs to match the producer side configs for metrics Renamed AUTO_COMMIT_ENABLE_CONFIG to ENABLE_AUTO_COMMIT_CONFIG Addressing review comments from Tim and Guozhang Rebasing after producer side config cleanup Added license headers Cleaned javadoc for ConsumerConfig Fixed minor indentation in ConsumerConfig Improve docs on ConsumerConfig 1. Added ClientUtils 2. Added basic constructor implementation for KafkaConsumer Improved MockConsumer Chris's feedback and also consumer rewind example code Added commit() and commitAsync() APIs to the consumer and updated docs and examples to reflect that 1. Added consumer usage examples to javadoc 2. Changed signature of APIs that accept or return offsets from list of offsets to map of offsets Improved example for using ConsumerRebalanceCallback Improved example for using ConsumerRebalanceCallback Included Jun's review comments and renamed positions to seek. Also included position() Changes to javadoc for positions() Changed the javadoc for ConsumerRebalanceCallback Changing unsubscribe to also take in var args for topic list Incorporated first round of feedback from Jay, Pradeep and Mattijs on the mailing list Updated configs Javadoc for consumer complete Completed docs for Consumer and ConsumerRebalanceCallback. Added MockConsumer Added the initial interfaces and related documentation for the consumer. More docs required to complete the public API Diffs - clients/src/main/java/org/apache/kafka/clients/consumer/Consumer.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerConfig.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRebalanceCallback.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRecord.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRecords.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/KafkaConsumer.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/MockConsumer.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/OffsetMetadata.java PRE-CREATION
Re: Review Request 19731: Patch for KAFKA-1328
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/19731/#review43385 --- clients/src/main/java/org/apache/kafka/clients/consumer/Consumer.java https://reviews.apache.org/r/19731/#comment77466 Should we discuss about code conventions of using ellipse (i.e. an array list) versus using Set, since Jun's reason here also applies to subscribe/unsubscribe? clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRecords.java https://reviews.apache.org/r/19731/#comment77468 I thought originally this class is used to wrap and expose the per-partition error code so that we do not need to throw an exception on every record of that partition. Is that still true? - Guozhang Wang On May 16, 2014, 6:46 p.m., Neha Narkhede wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/19731/ --- (Updated May 16, 2014, 6:46 p.m.) Review request for kafka. Bugs: KAFKA-1328 https://issues.apache.org/jira/browse/KAFKA-1328 Repository: kafka Description --- 1. Improved documentation on the position() API 2. Changed signature of commit API to remove Future and include a sync flag Included Jun's review suggestions part 2, except change to the commit() API since it needs more thought Review comments from Jun and Guozhang Checked in ConsumerRecordMetadata Fixed the javadoc usage examples in KafkaConsumer to match the API changes Changed the signature of poll to return MapString,ConsumerRecordMetadata to organize the ConsumerRecords around topic and then optionally around partition. This will serve the group management as well as custom partition subscription use cases 1. Changed the signature of poll() to return MapString, ListConsumerRecord 2. Changed ConsumerRecord to throw an exception if an error is detected for the partition. For example, if a single large message is larger than the total memory just for that partition, we don't want poll() to throw an exception since that will affect the processing of the remaining partitions as well Fixed MockConsumer to make subscribe(topics) and subscribe(partitions) mutually exclusive Changed the package to org.apache.kafka.clients.consumer from kafka.clients.consumer Changed the package to org.apache.kafka.clients.consumer from kafka.clients.consumer 1. Removed the commitAsync() APIs 2. Changed the commit() APIs to return a Future Fixed configs to match the producer side configs for metrics Renamed AUTO_COMMIT_ENABLE_CONFIG to ENABLE_AUTO_COMMIT_CONFIG Addressing review comments from Tim and Guozhang Rebasing after producer side config cleanup Added license headers Cleaned javadoc for ConsumerConfig Fixed minor indentation in ConsumerConfig Improve docs on ConsumerConfig 1. Added ClientUtils 2. Added basic constructor implementation for KafkaConsumer Improved MockConsumer Chris's feedback and also consumer rewind example code Added commit() and commitAsync() APIs to the consumer and updated docs and examples to reflect that 1. Added consumer usage examples to javadoc 2. Changed signature of APIs that accept or return offsets from list of offsets to map of offsets Improved example for using ConsumerRebalanceCallback Improved example for using ConsumerRebalanceCallback Included Jun's review comments and renamed positions to seek. Also included position() Changes to javadoc for positions() Changed the javadoc for ConsumerRebalanceCallback Changing unsubscribe to also take in var args for topic list Incorporated first round of feedback from Jay, Pradeep and Mattijs on the mailing list Updated configs Javadoc for consumer complete Completed docs for Consumer and ConsumerRebalanceCallback. Added MockConsumer Added the initial interfaces and related documentation for the consumer. More docs required to complete the public API Diffs - clients/src/main/java/org/apache/kafka/clients/consumer/Consumer.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerConfig.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRebalanceCallback.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRecord.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRecords.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/KafkaConsumer.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/MockConsumer.java PRE-CREATION
Re: Review Request 19731: Patch for KAFKA-1328
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/19731/#review43395 --- clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerConfig.java https://reviews.apache.org/r/19731/#comment77485 Have we added the config for selecting the strategy for assigning partitions? - Jun Rao On May 16, 2014, 6:46 p.m., Neha Narkhede wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/19731/ --- (Updated May 16, 2014, 6:46 p.m.) Review request for kafka. Bugs: KAFKA-1328 https://issues.apache.org/jira/browse/KAFKA-1328 Repository: kafka Description --- 1. Improved documentation on the position() API 2. Changed signature of commit API to remove Future and include a sync flag Included Jun's review suggestions part 2, except change to the commit() API since it needs more thought Review comments from Jun and Guozhang Checked in ConsumerRecordMetadata Fixed the javadoc usage examples in KafkaConsumer to match the API changes Changed the signature of poll to return MapString,ConsumerRecordMetadata to organize the ConsumerRecords around topic and then optionally around partition. This will serve the group management as well as custom partition subscription use cases 1. Changed the signature of poll() to return MapString, ListConsumerRecord 2. Changed ConsumerRecord to throw an exception if an error is detected for the partition. For example, if a single large message is larger than the total memory just for that partition, we don't want poll() to throw an exception since that will affect the processing of the remaining partitions as well Fixed MockConsumer to make subscribe(topics) and subscribe(partitions) mutually exclusive Changed the package to org.apache.kafka.clients.consumer from kafka.clients.consumer Changed the package to org.apache.kafka.clients.consumer from kafka.clients.consumer 1. Removed the commitAsync() APIs 2. Changed the commit() APIs to return a Future Fixed configs to match the producer side configs for metrics Renamed AUTO_COMMIT_ENABLE_CONFIG to ENABLE_AUTO_COMMIT_CONFIG Addressing review comments from Tim and Guozhang Rebasing after producer side config cleanup Added license headers Cleaned javadoc for ConsumerConfig Fixed minor indentation in ConsumerConfig Improve docs on ConsumerConfig 1. Added ClientUtils 2. Added basic constructor implementation for KafkaConsumer Improved MockConsumer Chris's feedback and also consumer rewind example code Added commit() and commitAsync() APIs to the consumer and updated docs and examples to reflect that 1. Added consumer usage examples to javadoc 2. Changed signature of APIs that accept or return offsets from list of offsets to map of offsets Improved example for using ConsumerRebalanceCallback Improved example for using ConsumerRebalanceCallback Included Jun's review comments and renamed positions to seek. Also included position() Changes to javadoc for positions() Changed the javadoc for ConsumerRebalanceCallback Changing unsubscribe to also take in var args for topic list Incorporated first round of feedback from Jay, Pradeep and Mattijs on the mailing list Updated configs Javadoc for consumer complete Completed docs for Consumer and ConsumerRebalanceCallback. Added MockConsumer Added the initial interfaces and related documentation for the consumer. More docs required to complete the public API Diffs - clients/src/main/java/org/apache/kafka/clients/consumer/Consumer.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerConfig.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRebalanceCallback.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRecord.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRecords.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/KafkaConsumer.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/MockConsumer.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/OffsetMetadata.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/internals/FutureOffsetMetadata.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/producer/KafkaProducer.java 90cacbd8941b7c8f15d1417c821945c1ac1b4d00 clients/src/main/java/org/apache/kafka/common/utils/ClientUtils.java PRE-CREATION
Re: Review Request 19731: Patch for KAFKA-1328
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/19731/#review42405 --- The javadoc for KafkaConsumer is missing in http://people.apache.org/~nehanarkhede/kafka-0.9-consumer-javadoc/doc/ clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRecord.java https://reviews.apache.org/r/19731/#comment76172 partitionId can probably be just partition to be consistent with what's in ProducerRecord. clients/src/main/java/org/apache/kafka/clients/consumer/KafkaConsumer.java https://reviews.apache.org/r/19731/#comment76174 We can remove either now. clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRecordMetadata.java https://reviews.apache.org/r/19731/#comment76173 Could this just be named ConsumerRecords? clients/src/main/java/org/apache/kafka/clients/consumer/Consumer.java https://reviews.apache.org/r/19731/#comment76175 The Future thing doesn't work well in this case. This is because the caller thread is also the one that does the polling. If the caller calls future.get, it will block forever since there won't be any polling so that we can get the response. So, we will likely have to make a separate blocking api. clients/src/main/java/org/apache/kafka/clients/consumer/Consumer.java https://reviews.apache.org/r/19731/#comment76176 If we can't think of a usage of this api, perhaps we should just remove it. The typically usage is that we want to seek to a previously committed offset. However, knowing the current fetch offset is of little use. clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRecord.java https://reviews.apache.org/r/19731/#comment76179 Perhaps RuntimeException should just be Exception to be consistent with the Callback api in the producer. clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRecord.java https://reviews.apache.org/r/19731/#comment76180 Perhaps we just need to combine the two into one api topicAndPartition(). clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRecord.java https://reviews.apache.org/r/19731/#comment76178 The issue with not exposing a nextOffset() api is that users have to figure out the next offset themselves, which is not natural. The common usage is the app finishes consuming a record and want to commit the next offset (not the current offset) after the consumed record. Having a nextOffset() will allow us to explain this to the user better in the api. clients/src/main/java/org/apache/kafka/clients/consumer/KafkaConsumer.java https://reviews.apache.org/r/19731/#comment76177 The returned offset should be the next offset. See the comment on exposing nextOffset(). - Jun Rao On May 5, 2014, 6:35 p.m., Neha Narkhede wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/19731/ --- (Updated May 5, 2014, 6:35 p.m.) Review request for kafka. Bugs: KAFKA-1328 https://issues.apache.org/jira/browse/KAFKA-1328 Repository: kafka Description --- Review comments from Jun and Guozhang Checked in ConsumerRecordMetadata Fixed the javadoc usage examples in KafkaConsumer to match the API changes Changed the signature of poll to return MapString,ConsumerRecordMetadata to organize the ConsumerRecords around topic and then optionally around partition. This will serve the group management as well as custom partition subscription use cases 1. Changed the signature of poll() to return MapString, ListConsumerRecord 2. Changed ConsumerRecord to throw an exception if an error is detected for the partition. For example, if a single large message is larger than the total memory just for that partition, we don't want poll() to throw an exception since that will affect the processing of the remaining partitions as well Fixed MockConsumer to make subscribe(topics) and subscribe(partitions) mutually exclusive Changed the package to org.apache.kafka.clients.consumer from kafka.clients.consumer Changed the package to org.apache.kafka.clients.consumer from kafka.clients.consumer 1. Removed the commitAsync() APIs 2. Changed the commit() APIs to return a Future Fixed configs to match the producer side configs for metrics Renamed AUTO_COMMIT_ENABLE_CONFIG to ENABLE_AUTO_COMMIT_CONFIG Addressing review comments from Tim and Guozhang Rebasing after producer side config cleanup Added license headers Cleaned javadoc for ConsumerConfig Fixed minor indentation in ConsumerConfig Improve docs on ConsumerConfig 1. Added ClientUtils 2. Added basic constructor implementation for KafkaConsumer Improved
Re: Review Request 19731: Patch for KAFKA-1328
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/19731/ --- (Updated May 10, 2014, 12:18 a.m.) Review request for kafka. Bugs: KAFKA-1328 https://issues.apache.org/jira/browse/KAFKA-1328 Repository: kafka Description (updated) --- Included Jun's review suggestions part 2, except change to the commit() API since it needs more thought Review comments from Jun and Guozhang Checked in ConsumerRecordMetadata Fixed the javadoc usage examples in KafkaConsumer to match the API changes Changed the signature of poll to return MapString,ConsumerRecordMetadata to organize the ConsumerRecords around topic and then optionally around partition. This will serve the group management as well as custom partition subscription use cases 1. Changed the signature of poll() to return MapString, ListConsumerRecord 2. Changed ConsumerRecord to throw an exception if an error is detected for the partition. For example, if a single large message is larger than the total memory just for that partition, we don't want poll() to throw an exception since that will affect the processing of the remaining partitions as well Fixed MockConsumer to make subscribe(topics) and subscribe(partitions) mutually exclusive Changed the package to org.apache.kafka.clients.consumer from kafka.clients.consumer Changed the package to org.apache.kafka.clients.consumer from kafka.clients.consumer 1. Removed the commitAsync() APIs 2. Changed the commit() APIs to return a Future Fixed configs to match the producer side configs for metrics Renamed AUTO_COMMIT_ENABLE_CONFIG to ENABLE_AUTO_COMMIT_CONFIG Addressing review comments from Tim and Guozhang Rebasing after producer side config cleanup Added license headers Cleaned javadoc for ConsumerConfig Fixed minor indentation in ConsumerConfig Improve docs on ConsumerConfig 1. Added ClientUtils 2. Added basic constructor implementation for KafkaConsumer Improved MockConsumer Chris's feedback and also consumer rewind example code Added commit() and commitAsync() APIs to the consumer and updated docs and examples to reflect that 1. Added consumer usage examples to javadoc 2. Changed signature of APIs that accept or return offsets from list of offsets to map of offsets Improved example for using ConsumerRebalanceCallback Improved example for using ConsumerRebalanceCallback Included Jun's review comments and renamed positions to seek. Also included position() Changes to javadoc for positions() Changed the javadoc for ConsumerRebalanceCallback Changing unsubscribe to also take in var args for topic list Incorporated first round of feedback from Jay, Pradeep and Mattijs on the mailing list Updated configs Javadoc for consumer complete Completed docs for Consumer and ConsumerRebalanceCallback. Added MockConsumer Added the initial interfaces and related documentation for the consumer. More docs required to complete the public API Diffs (updated) - clients/src/main/java/org/apache/kafka/clients/consumer/Consumer.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerConfig.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRebalanceCallback.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRecord.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRecords.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/KafkaConsumer.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/MockConsumer.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/OffsetMetadata.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/internals/FutureOffsetMetadata.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/producer/KafkaProducer.java a6423f4b37a57f0290e2048b764de1218470f4f7 clients/src/main/java/org/apache/kafka/common/utils/ClientUtils.java PRE-CREATION clients/src/test/java/org/apache/kafka/clients/consumer/ConsumerExampleTest.java PRE-CREATION Diff: https://reviews.apache.org/r/19731/diff/ Testing --- Thanks, Neha Narkhede
Re: Review Request 19731: Patch for KAFKA-1328
On May 7, 2014, 4:50 p.m., Jun Rao wrote: clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRecord.java, lines 37-38 https://reviews.apache.org/r/19731/diff/7-8/?file=555628#file555628line37 partitionId can probably be just partition to be consistent with what's in ProducerRecord. It's probably going to require a change on the producer. See my comment on this rb previously - The returned object from partition() is TopicPartition on purpose. I realized that returning partition id from this API is useless since all other APIs in the consumer accept TopicPartition. The constructor parameter can be renamed to partitionId On May 7, 2014, 4:50 p.m., Jun Rao wrote: clients/src/main/java/org/apache/kafka/clients/consumer/Consumer.java, lines 98-104 https://reviews.apache.org/r/19731/diff/9/?file=574802#file574802line98 If we can't think of a usage of this api, perhaps we should just remove it. The typically usage is that we want to seek to a previously committed offset. However, knowing the current fetch offset is of little use. Please refer to the mailing list discussion on the requirement for this API. The subject was something like New consumer API discussion. On May 7, 2014, 4:50 p.m., Jun Rao wrote: clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRecord.java, lines 108-109 https://reviews.apache.org/r/19731/diff/9/?file=574805#file574805line108 The issue with not exposing a nextOffset() api is that users have to figure out the next offset themselves, which is not natural. The common usage is the app finishes consuming a record and want to commit the next offset (not the current offset) after the consumed record. Having a nextOffset() will allow us to explain this to the user better in the api. Let's discuss this more explicitly on the mailing list. I'm not opposed to exposing the API if most people feel the need for it. Make sure you give this feedback on the API discussion thread as well. I can make this change after the initial patch is in. On May 7, 2014, 4:50 p.m., Jun Rao wrote: clients/src/main/java/org/apache/kafka/clients/consumer/KafkaConsumer.java, lines 53-63 https://reviews.apache.org/r/19731/diff/9/?file=574807#file574807line53 The returned offset should be the next offset. See the comment on exposing nextOffset(). Not really. processedOffsets here stores offsets of records for which the consumer has finished processing. So record.offset() seems correct right? On May 7, 2014, 4:50 p.m., Jun Rao wrote: clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRecord.java, lines 74-86 https://reviews.apache.org/r/19731/diff/9/?file=574805#file574805line74 Perhaps we just need to combine the two into one api topicAndPartition(). We could definitely add a topicAndPartition() API. On May 7, 2014, 4:50 p.m., Jun Rao wrote: clients/src/main/java/org/apache/kafka/clients/consumer/Consumer.java, lines 87-88 https://reviews.apache.org/r/19731/diff/9/?file=574802#file574802line87 The Future thing doesn't work well in this case. This is because the caller thread is also the one that does the polling. If the caller calls future.get, it will block forever since there won't be any polling so that we can get the response. So, we will likely have to make a separate blocking api. Hmm.. even if we expose it as a separate API, it seems the problem you mentioned will not go away. Probably adding a callback is a better approach. - Neha --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/19731/#review42405 --- On May 5, 2014, 6:35 p.m., Neha Narkhede wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/19731/ --- (Updated May 5, 2014, 6:35 p.m.) Review request for kafka. Bugs: KAFKA-1328 https://issues.apache.org/jira/browse/KAFKA-1328 Repository: kafka Description --- Review comments from Jun and Guozhang Checked in ConsumerRecordMetadata Fixed the javadoc usage examples in KafkaConsumer to match the API changes Changed the signature of poll to return MapString,ConsumerRecordMetadata to organize the ConsumerRecords around topic and then optionally around partition. This will serve the group management as well as custom partition subscription use cases 1. Changed the signature of poll() to return MapString, ListConsumerRecord 2. Changed ConsumerRecord to throw an exception if an error is detected for the partition. For example, if a single large message is larger than the total memory just for that partition, we don't want poll() to
Re: Review Request 19731: Patch for KAFKA-1328
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/19731/#review42787 --- clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerConfig.java https://reviews.apache.org/r/19731/#comment76701 Do we need a config that allows the consumer to pick a strategy for dividing partitions among consumers? - Jun Rao On May 10, 2014, 12:18 a.m., Neha Narkhede wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/19731/ --- (Updated May 10, 2014, 12:18 a.m.) Review request for kafka. Bugs: KAFKA-1328 https://issues.apache.org/jira/browse/KAFKA-1328 Repository: kafka Description --- Included Jun's review suggestions part 2, except change to the commit() API since it needs more thought Review comments from Jun and Guozhang Checked in ConsumerRecordMetadata Fixed the javadoc usage examples in KafkaConsumer to match the API changes Changed the signature of poll to return MapString,ConsumerRecordMetadata to organize the ConsumerRecords around topic and then optionally around partition. This will serve the group management as well as custom partition subscription use cases 1. Changed the signature of poll() to return MapString, ListConsumerRecord 2. Changed ConsumerRecord to throw an exception if an error is detected for the partition. For example, if a single large message is larger than the total memory just for that partition, we don't want poll() to throw an exception since that will affect the processing of the remaining partitions as well Fixed MockConsumer to make subscribe(topics) and subscribe(partitions) mutually exclusive Changed the package to org.apache.kafka.clients.consumer from kafka.clients.consumer Changed the package to org.apache.kafka.clients.consumer from kafka.clients.consumer 1. Removed the commitAsync() APIs 2. Changed the commit() APIs to return a Future Fixed configs to match the producer side configs for metrics Renamed AUTO_COMMIT_ENABLE_CONFIG to ENABLE_AUTO_COMMIT_CONFIG Addressing review comments from Tim and Guozhang Rebasing after producer side config cleanup Added license headers Cleaned javadoc for ConsumerConfig Fixed minor indentation in ConsumerConfig Improve docs on ConsumerConfig 1. Added ClientUtils 2. Added basic constructor implementation for KafkaConsumer Improved MockConsumer Chris's feedback and also consumer rewind example code Added commit() and commitAsync() APIs to the consumer and updated docs and examples to reflect that 1. Added consumer usage examples to javadoc 2. Changed signature of APIs that accept or return offsets from list of offsets to map of offsets Improved example for using ConsumerRebalanceCallback Improved example for using ConsumerRebalanceCallback Included Jun's review comments and renamed positions to seek. Also included position() Changes to javadoc for positions() Changed the javadoc for ConsumerRebalanceCallback Changing unsubscribe to also take in var args for topic list Incorporated first round of feedback from Jay, Pradeep and Mattijs on the mailing list Updated configs Javadoc for consumer complete Completed docs for Consumer and ConsumerRebalanceCallback. Added MockConsumer Added the initial interfaces and related documentation for the consumer. More docs required to complete the public API Diffs - clients/src/main/java/org/apache/kafka/clients/consumer/Consumer.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerConfig.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRebalanceCallback.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRecord.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRecords.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/KafkaConsumer.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/MockConsumer.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/OffsetMetadata.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/internals/FutureOffsetMetadata.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/producer/KafkaProducer.java a6423f4b37a57f0290e2048b764de1218470f4f7 clients/src/main/java/org/apache/kafka/common/utils/ClientUtils.java PRE-CREATION clients/src/test/java/org/apache/kafka/clients/consumer/ConsumerExampleTest.java PRE-CREATION
Re: Review Request 19731: Patch for KAFKA-1328
On May 7, 2014, 4:50 p.m., Jun Rao wrote: clients/src/main/java/org/apache/kafka/clients/consumer/Consumer.java, lines 87-88 https://reviews.apache.org/r/19731/diff/9/?file=574802#file574802line87 The Future thing doesn't work well in this case. This is because the caller thread is also the one that does the polling. If the caller calls future.get, it will block forever since there won't be any polling so that we can get the response. So, we will likely have to make a separate blocking api. Neha Narkhede wrote: Hmm.. even if we expose it as a separate API, it seems the problem you mentioned will not go away. Probably adding a callback is a better approach. I was thinking that the implementation of the syncCommit() call will just keep calling selector.select() until the commit response is received. On May 7, 2014, 4:50 p.m., Jun Rao wrote: clients/src/main/java/org/apache/kafka/clients/consumer/KafkaConsumer.java, lines 53-63 https://reviews.apache.org/r/19731/diff/9/?file=574807#file574807line53 The returned offset should be the next offset. See the comment on exposing nextOffset(). Neha Narkhede wrote: Not really. processedOffsets here stores offsets of records for which the consumer has finished processing. So record.offset() seems correct right? If an app completes the consumption of a message m and then calls commit, the expectation is that if the app fails immediately after commit, it will resume consumption from the next message after m, since m has been previously consumed successfully. - Jun --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/19731/#review42405 --- On May 10, 2014, 12:18 a.m., Neha Narkhede wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/19731/ --- (Updated May 10, 2014, 12:18 a.m.) Review request for kafka. Bugs: KAFKA-1328 https://issues.apache.org/jira/browse/KAFKA-1328 Repository: kafka Description --- Included Jun's review suggestions part 2, except change to the commit() API since it needs more thought Review comments from Jun and Guozhang Checked in ConsumerRecordMetadata Fixed the javadoc usage examples in KafkaConsumer to match the API changes Changed the signature of poll to return MapString,ConsumerRecordMetadata to organize the ConsumerRecords around topic and then optionally around partition. This will serve the group management as well as custom partition subscription use cases 1. Changed the signature of poll() to return MapString, ListConsumerRecord 2. Changed ConsumerRecord to throw an exception if an error is detected for the partition. For example, if a single large message is larger than the total memory just for that partition, we don't want poll() to throw an exception since that will affect the processing of the remaining partitions as well Fixed MockConsumer to make subscribe(topics) and subscribe(partitions) mutually exclusive Changed the package to org.apache.kafka.clients.consumer from kafka.clients.consumer Changed the package to org.apache.kafka.clients.consumer from kafka.clients.consumer 1. Removed the commitAsync() APIs 2. Changed the commit() APIs to return a Future Fixed configs to match the producer side configs for metrics Renamed AUTO_COMMIT_ENABLE_CONFIG to ENABLE_AUTO_COMMIT_CONFIG Addressing review comments from Tim and Guozhang Rebasing after producer side config cleanup Added license headers Cleaned javadoc for ConsumerConfig Fixed minor indentation in ConsumerConfig Improve docs on ConsumerConfig 1. Added ClientUtils 2. Added basic constructor implementation for KafkaConsumer Improved MockConsumer Chris's feedback and also consumer rewind example code Added commit() and commitAsync() APIs to the consumer and updated docs and examples to reflect that 1. Added consumer usage examples to javadoc 2. Changed signature of APIs that accept or return offsets from list of offsets to map of offsets Improved example for using ConsumerRebalanceCallback Improved example for using ConsumerRebalanceCallback Included Jun's review comments and renamed positions to seek. Also included position() Changes to javadoc for positions() Changed the javadoc for ConsumerRebalanceCallback Changing unsubscribe to also take in var args for topic list Incorporated first round of feedback from Jay, Pradeep and Mattijs on the mailing list Updated configs Javadoc for consumer complete Completed docs for Consumer and ConsumerRebalanceCallback.
Re: Review Request 19731: Patch for KAFKA-1328
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/19731/#review41974 --- clients/src/main/java/org/apache/kafka/clients/consumer/Consumer.java https://reviews.apache.org/r/19731/#comment75659 Did you mean unchecked exception? If so, yes. This is consistent with the producer, I think. clients/src/main/java/org/apache/kafka/clients/consumer/Consumer.java https://reviews.apache.org/r/19731/#comment75669 Right. I started with a simple list of ConsumerRecord but found it very painful while writing the examples. Basically whether or not clients prefer collation by topic or partition depends on the nature of the use case, threadpool processing strategy as well as topic vs partition subscription. Another thing is figuring out how to throw a per partition exception. This led to an API design that allows collation by topic as well as partition. If we need to throw partition level exceptions, ConsumerRecordMetadata would be one way of exposing those clients/src/main/java/org/apache/kafka/clients/consumer/Consumer.java https://reviews.apache.org/r/19731/#comment75670 Yes. If the user is not interested in some partitions, it is best to unsubscribe. commit() will always commit offsets for all subscribed partitions owned by the consumer. clients/src/main/java/org/apache/kafka/clients/consumer/Consumer.java https://reviews.apache.org/r/19731/#comment75790 Yes, I thought of that, but changed it to a separate OffsetMetadata object for a couple of reasons 1. It provides flexibility to allow us to expose more information, if required. For example, return the last committed offset if the commit for a particular partition fails. 2. It stays consistent with the producer client APIs where we don't return the error code value to the user but instead throw an exception while accessing the data (in this case the offset. However, I see your point about returning complex data back. I think it can be simplified by returning a Future of OffsetMetadata and changing OffsetMetadata to have an offset() API that returns either the last committed offset or throws an exception. That might address your concern as well as get us the flexibility and consistency. Thoughts? clients/src/main/java/org/apache/kafka/clients/consumer/Consumer.java https://reviews.apache.org/r/19731/#comment75885 Yes, that is better for consistency clients/src/main/java/org/apache/kafka/clients/consumer/Consumer.java https://reviews.apache.org/r/19731/#comment75886 This is already a batch API. clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerConfig.java https://reviews.apache.org/r/19731/#comment75896 auto.offset.reset=disable will expect the consumer to set the offset before the first poll(). This can be done using commit() and seek(). clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerConfig.java https://reviews.apache.org/r/19731/#comment75898 The memory management on the consumer is going to require server side changes. For example, if the consumer's fetch request contains a max limit (set using total.memory.bytes) and a fetch.buffer.bytes, the server will return at least fetch.buffer.bytes from a subset of the n partitions. The server selects the partitions in round robin or randomly. We can discuss more details in the design review. clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerConfig.java https://reviews.apache.org/r/19731/#comment75899 Removed clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRecord.java https://reviews.apache.org/r/19731/#comment75900 The returned object from partition() is TopicPartition on purpose. I realized that returning partition id from this API is useless since all other APIs in the consumer accept TopicPartition. The constructor parameter can be renamed to partitionId. clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRecord.java https://reviews.apache.org/r/19731/#comment75901 Should we? It is a little odd that the returned record has an API called nextOffset(), especially since we are moving away from an iterator like API to a collection of records kind API. The downside ofcourse is that we are exposing the assumption that the offset of the next available message is currentOffset+1. However, I would argue that it is the most logical expected behavior from Kafka that we should never change. clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRecord.java https://reviews.apache.org/r/19731/#comment75902 This is to keep it consistent with the producer side. Also, I think from a user perspective an exception is more intuitive and an integer error code, no? clients/src/main/java/org/apache/kafka/clients/consumer/KafkaConsumer.java
Re: Review Request 19731: Patch for KAFKA-1328
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/19731/ --- (Updated May 5, 2014, 6:35 p.m.) Review request for kafka. Bugs: KAFKA-1328 https://issues.apache.org/jira/browse/KAFKA-1328 Repository: kafka Description (updated) --- Review comments from Jun and Guozhang Checked in ConsumerRecordMetadata Fixed the javadoc usage examples in KafkaConsumer to match the API changes Changed the signature of poll to return MapString,ConsumerRecordMetadata to organize the ConsumerRecords around topic and then optionally around partition. This will serve the group management as well as custom partition subscription use cases 1. Changed the signature of poll() to return MapString, ListConsumerRecord 2. Changed ConsumerRecord to throw an exception if an error is detected for the partition. For example, if a single large message is larger than the total memory just for that partition, we don't want poll() to throw an exception since that will affect the processing of the remaining partitions as well Fixed MockConsumer to make subscribe(topics) and subscribe(partitions) mutually exclusive Changed the package to org.apache.kafka.clients.consumer from kafka.clients.consumer Changed the package to org.apache.kafka.clients.consumer from kafka.clients.consumer 1. Removed the commitAsync() APIs 2. Changed the commit() APIs to return a Future Fixed configs to match the producer side configs for metrics Renamed AUTO_COMMIT_ENABLE_CONFIG to ENABLE_AUTO_COMMIT_CONFIG Addressing review comments from Tim and Guozhang Rebasing after producer side config cleanup Added license headers Cleaned javadoc for ConsumerConfig Fixed minor indentation in ConsumerConfig Improve docs on ConsumerConfig 1. Added ClientUtils 2. Added basic constructor implementation for KafkaConsumer Improved MockConsumer Chris's feedback and also consumer rewind example code Added commit() and commitAsync() APIs to the consumer and updated docs and examples to reflect that 1. Added consumer usage examples to javadoc 2. Changed signature of APIs that accept or return offsets from list of offsets to map of offsets Improved example for using ConsumerRebalanceCallback Improved example for using ConsumerRebalanceCallback Included Jun's review comments and renamed positions to seek. Also included position() Changes to javadoc for positions() Changed the javadoc for ConsumerRebalanceCallback Changing unsubscribe to also take in var args for topic list Incorporated first round of feedback from Jay, Pradeep and Mattijs on the mailing list Updated configs Javadoc for consumer complete Completed docs for Consumer and ConsumerRebalanceCallback. Added MockConsumer Added the initial interfaces and related documentation for the consumer. More docs required to complete the public API Diffs (updated) - clients/src/main/java/org/apache/kafka/clients/consumer/Consumer.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerConfig.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRebalanceCallback.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRecord.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRecordMetadata.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/KafkaConsumer.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/MockConsumer.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/OffsetMetadata.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/internals/FutureOffsetMetadata.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/producer/KafkaProducer.java a6423f4b37a57f0290e2048b764de1218470f4f7 clients/src/main/java/org/apache/kafka/common/utils/ClientUtils.java PRE-CREATION clients/src/test/java/org/apache/kafka/clients/consumer/ConsumerExampleTest.java PRE-CREATION Diff: https://reviews.apache.org/r/19731/diff/ Testing --- Thanks, Neha Narkhede
Re: Review Request 19731: Patch for KAFKA-1328
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/19731/ --- (Updated May 5, 2014, 6:35 p.m.) Review request for kafka. Bugs: KAFKA-1328 https://issues.apache.org/jira/browse/KAFKA-1328 Repository: kafka Description --- Review comments from Jun and Guozhang Checked in ConsumerRecordMetadata Fixed the javadoc usage examples in KafkaConsumer to match the API changes Changed the signature of poll to return MapString,ConsumerRecordMetadata to organize the ConsumerRecords around topic and then optionally around partition. This will serve the group management as well as custom partition subscription use cases 1. Changed the signature of poll() to return MapString, ListConsumerRecord 2. Changed ConsumerRecord to throw an exception if an error is detected for the partition. For example, if a single large message is larger than the total memory just for that partition, we don't want poll() to throw an exception since that will affect the processing of the remaining partitions as well Fixed MockConsumer to make subscribe(topics) and subscribe(partitions) mutually exclusive Changed the package to org.apache.kafka.clients.consumer from kafka.clients.consumer Changed the package to org.apache.kafka.clients.consumer from kafka.clients.consumer 1. Removed the commitAsync() APIs 2. Changed the commit() APIs to return a Future Fixed configs to match the producer side configs for metrics Renamed AUTO_COMMIT_ENABLE_CONFIG to ENABLE_AUTO_COMMIT_CONFIG Addressing review comments from Tim and Guozhang Rebasing after producer side config cleanup Added license headers Cleaned javadoc for ConsumerConfig Fixed minor indentation in ConsumerConfig Improve docs on ConsumerConfig 1. Added ClientUtils 2. Added basic constructor implementation for KafkaConsumer Improved MockConsumer Chris's feedback and also consumer rewind example code Added commit() and commitAsync() APIs to the consumer and updated docs and examples to reflect that 1. Added consumer usage examples to javadoc 2. Changed signature of APIs that accept or return offsets from list of offsets to map of offsets Improved example for using ConsumerRebalanceCallback Improved example for using ConsumerRebalanceCallback Included Jun's review comments and renamed positions to seek. Also included position() Changes to javadoc for positions() Changed the javadoc for ConsumerRebalanceCallback Changing unsubscribe to also take in var args for topic list Incorporated first round of feedback from Jay, Pradeep and Mattijs on the mailing list Updated configs Javadoc for consumer complete Completed docs for Consumer and ConsumerRebalanceCallback. Added MockConsumer Added the initial interfaces and related documentation for the consumer. More docs required to complete the public API Diffs (updated) - clients/src/main/java/org/apache/kafka/clients/consumer/Consumer.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerConfig.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRebalanceCallback.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRecord.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRecordMetadata.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/KafkaConsumer.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/MockConsumer.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/OffsetMetadata.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/internals/FutureOffsetMetadata.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/producer/KafkaProducer.java a6423f4b37a57f0290e2048b764de1218470f4f7 clients/src/main/java/org/apache/kafka/common/utils/ClientUtils.java PRE-CREATION clients/src/test/java/org/apache/kafka/clients/consumer/ConsumerExampleTest.java PRE-CREATION Diff: https://reviews.apache.org/r/19731/diff/ Testing --- Thanks, Neha Narkhede
Re: Review Request 19731: Patch for KAFKA-1328
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/19731/#review40856 --- 1. I am not sure about the use of ellipsis. This may make passing a set of items from a collection to the api a bit harder. Suppose that you have a list of topics stored in ArrayListString topics; If you want subscribe to all topics in one call, you will have to do: String[] topicArray = new String[topics.size()]; consumer.subscribe(topics.toArray(topicArray)); See the comment on the example below. Also, passing in an ellipsis and getting back a map is a bit weird. Passing in ellipsis doesn't make it obvious that partitions from the input should be unique. It probably would be more natural if we pass in a Set instead. clients/src/main/java/kafka/common/TopicPartitionOffset.java https://reviews.apache.org/r/19731/#comment75074 Apache license header. clients/src/main/java/org/apache/kafka/clients/consumer/Consumer.java https://reviews.apache.org/r/19731/#comment75057 What's the error when unsubscribing a wrong topic/partition? Do we throw an uncaught exception? clients/src/main/java/org/apache/kafka/clients/consumer/Consumer.java https://reviews.apache.org/r/19731/#comment75058 Some apps want to collate by topic and some others may want to collate by partition. It seems it's simpler to just return a list of ConsumerRecord. Another alternative is to return a partition, record map which seems to match the rest of the apis better. clients/src/main/java/org/apache/kafka/clients/consumer/Consumer.java https://reviews.apache.org/r/19731/#comment74106 Each poll may not return all subscribed partitions. So, when calling commit(), do we expect to commit the offset of the last record for each subscribed partition returned by all previous polls? clients/src/main/java/org/apache/kafka/clients/consumer/Consumer.java https://reviews.apache.org/r/19731/#comment75064 Since we have a committed api, the return offset value in commit() seems redundant. Perhaps it's cleaner to just return FutureMapTopicPartition, errorCode? clients/src/main/java/org/apache/kafka/clients/consumer/Consumer.java https://reviews.apache.org/r/19731/#comment75062 We should make it clear that calling get() on the future will block until the commit request completes. clients/src/main/java/org/apache/kafka/clients/consumer/Consumer.java https://reviews.apache.org/r/19731/#comment75063 Should we make this a batch api to match seek()? clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerConfig.java https://reviews.apache.org/r/19731/#comment75072 It's probably reasonable to reset the offset based on an arbitrary timestamp using the getOffsetBefore() api. The granularity is coarse right now, but can be improved in the future. clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerConfig.java https://reviews.apache.org/r/19731/#comment75073 Do we need this and metadata.fetch.backoff.ms? clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRecord.java https://reviews.apache.org/r/19731/#comment75060 Should we also have an api for nextOffset()? clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRecord.java https://reviews.apache.org/r/19731/#comment75059 It seems it's more natural to return an error code than an exception. clients/src/main/java/org/apache/kafka/clients/consumer/KafkaConsumer.java https://reviews.apache.org/r/19731/#comment75071 The description on threading is a bit confusing. KafkaConsumer doesn't own any thread. All calls are done in the caller thread. It seems the easiest way to use the api is to call KafkaConsumer api from a single thread and pass the consumed records to a separate thread pool for parallel consumption if needed. clients/src/main/java/org/apache/kafka/clients/consumer/KafkaConsumer.java https://reviews.apache.org/r/19731/#comment75066 commitAsync() no longer exists. clients/src/main/java/org/apache/kafka/clients/consumer/KafkaConsumer.java https://reviews.apache.org/r/19731/#comment75067 The most common use case would be that the app wants auto rebalancing, but wants to commit offsets manually. So, we need to call commit() on partitionRevoked() and call seek with the committed offset in partitionAssigned(). This is actually the default behavior of the ConsumerRebalanceCallback. So, I am not sure if this example covers a common use case. clients/src/main/java/org/apache/kafka/clients/consumer/KafkaConsumer.java https://reviews.apache.org/r/19731/#comment75068 See the comment on ellipsis. Passing in partitions as ellipsis is a bit tricky. clients/src/main/java/org/apache/kafka/clients/consumer/KafkaConsumer.java https://reviews.apache.org/r/19731/#comment75069 We should commit the offset before
Re: Review Request 19731: Patch for KAFKA-1328
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/19731/#review40262 --- Some general comments: 1. Do we need to take parameters for position and committed? Shall we just let these two functions return offsets of the owned partitions, and users could just try to access the partitions of interest? 2. The usage of seek in the try-catch example seems a little awkward to me: do we really need to re-set the fetching position in processing failures? If people are afraid that this would be the case, should they be processing message one-at-a-time, like: for (record in records) { try { process(record) } catch { // resume and re-process(record)... } } clients/src/main/java/org/apache/kafka/clients/consumer/Consumer.java https://reviews.apache.org/r/19731/#comment73213 I think if some TopicPartition is not currently owned by the consumer, an exception will be thrown? If yes shall we state that in the comments? clients/src/main/java/org/apache/kafka/clients/consumer/Consumer.java https://reviews.apache.org/r/19731/#comment73214 Same as above? clients/src/main/java/org/apache/kafka/clients/consumer/Consumer.java https://reviews.apache.org/r/19731/#comment73215 Same as above? clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerConfig.java https://reviews.apache.org/r/19731/#comment73211 In the examples this config string is still auto.commit.enable clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerConfig.java https://reviews.apache.org/r/19731/#comment73216 I am not sure how we can do strict memory management at the consumer, since: 1) we do not have sth. like a fetch.max.bytes to upper-bound the returned response for fetch requests. So we can theoretically get a response that is as large as socket.receive.buffer.bytes; 2) when we read from the socket for responses, we have to read it as a whole and put it into a single MemoryRecords object. Only when we will know the size of the response. clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRecord.java https://reviews.apache.org/r/19731/#comment73427 There is a discrepancy in partition: in the constructor parameters it is the int partition id, in the getter function it refers to the TopicPartition. Perhaps we can rename the constructor parameters to partitionId, and add a getter function public int partitionId(). clients/src/main/java/org/apache/kafka/clients/consumer/KafkaConsumer.java https://reviews.apache.org/r/19731/#comment73433 When need to change the examples with the new signature of poll/commit/etc once we have finalized the API. clients/src/main/java/org/apache/kafka/clients/producer/KafkaProducer.java https://reviews.apache.org/r/19731/#comment73434 Rebase? clients/src/main/java/org/apache/kafka/common/utils/ClientUtils.java https://reviews.apache.org/r/19731/#comment73435 Ditto as above - Guozhang Wang On April 13, 2014, 2:12 a.m., Neha Narkhede wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/19731/ --- (Updated April 13, 2014, 2:12 a.m.) Review request for kafka. Bugs: KAFKA-1328 https://issues.apache.org/jira/browse/KAFKA-1328 Repository: kafka Description --- Fixed the javadoc usage examples in KafkaConsumer to match the API changes Changed the signature of poll to return MapString,ConsumerRecordMetadata to organize the ConsumerRecords around topic and then optionally around partition. This will serve the group management as well as custom partition subscription use cases 1. Changed the signature of poll() to return MapString, ListConsumerRecord 2. Changed ConsumerRecord to throw an exception if an error is detected for the partition. For example, if a single large message is larger than the total memory just for that partition, we don't want poll() to throw an exception since that will affect the processing of the remaining partitions as well Fixed MockConsumer to make subscribe(topics) and subscribe(partitions) mutually exclusive Changed the package to org.apache.kafka.clients.consumer from kafka.clients.consumer Changed the package to org.apache.kafka.clients.consumer from kafka.clients.consumer 1. Removed the commitAsync() APIs 2. Changed the commit() APIs to return a Future Fixed configs to match the producer side configs for metrics Renamed AUTO_COMMIT_ENABLE_CONFIG to ENABLE_AUTO_COMMIT_CONFIG Addressing review comments from Tim and Guozhang Rebasing after producer side config cleanup Added license headers Cleaned javadoc for ConsumerConfig
Re: Review Request 19731: Patch for KAFKA-1328
On April 7, 2014, 6:03 p.m., Guozhang Wang wrote: clients/src/main/java/kafka/clients/consumer/ConsumerRecord.java, line 22 https://reviews.apache.org/r/19731/diff/1/?file=538463#file538463line22 Suggest add the following functions: compressionType() Neha Narkhede wrote: I'm not so sure that it will make sense. The consumer record is always the original raw record and is always handed out after the decompression anyways. Thoughts? As we have discussed off-line, since we are always going to return the raw messages, i.e. use deep iterators, I think we do not need this attribute. - Guozhang --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/19731/#review39700 --- On April 13, 2014, 2:12 a.m., Neha Narkhede wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/19731/ --- (Updated April 13, 2014, 2:12 a.m.) Review request for kafka. Bugs: KAFKA-1328 https://issues.apache.org/jira/browse/KAFKA-1328 Repository: kafka Description --- Fixed the javadoc usage examples in KafkaConsumer to match the API changes Changed the signature of poll to return MapString,ConsumerRecordMetadata to organize the ConsumerRecords around topic and then optionally around partition. This will serve the group management as well as custom partition subscription use cases 1. Changed the signature of poll() to return MapString, ListConsumerRecord 2. Changed ConsumerRecord to throw an exception if an error is detected for the partition. For example, if a single large message is larger than the total memory just for that partition, we don't want poll() to throw an exception since that will affect the processing of the remaining partitions as well Fixed MockConsumer to make subscribe(topics) and subscribe(partitions) mutually exclusive Changed the package to org.apache.kafka.clients.consumer from kafka.clients.consumer Changed the package to org.apache.kafka.clients.consumer from kafka.clients.consumer 1. Removed the commitAsync() APIs 2. Changed the commit() APIs to return a Future Fixed configs to match the producer side configs for metrics Renamed AUTO_COMMIT_ENABLE_CONFIG to ENABLE_AUTO_COMMIT_CONFIG Addressing review comments from Tim and Guozhang Rebasing after producer side config cleanup Added license headers Cleaned javadoc for ConsumerConfig Fixed minor indentation in ConsumerConfig Improve docs on ConsumerConfig 1. Added ClientUtils 2. Added basic constructor implementation for KafkaConsumer Improved MockConsumer Chris's feedback and also consumer rewind example code Added commit() and commitAsync() APIs to the consumer and updated docs and examples to reflect that 1. Added consumer usage examples to javadoc 2. Changed signature of APIs that accept or return offsets from list of offsets to map of offsets Improved example for using ConsumerRebalanceCallback Improved example for using ConsumerRebalanceCallback Included Jun's review comments and renamed positions to seek. Also included position() Changes to javadoc for positions() Changed the javadoc for ConsumerRebalanceCallback Changing unsubscribe to also take in var args for topic list Incorporated first round of feedback from Jay, Pradeep and Mattijs on the mailing list Updated configs Javadoc for consumer complete Completed docs for Consumer and ConsumerRebalanceCallback. Added MockConsumer Added the initial interfaces and related documentation for the consumer. More docs required to complete the public API Diffs - clients/src/main/java/kafka/common/TopicPartitionOffset.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/Consumer.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerConfig.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRebalanceCallback.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRecord.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/KafkaConsumer.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/MockConsumer.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/OffsetMetadata.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/internals/FutureOffsetMetadata.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/producer/KafkaProducer.java a6423f4b37a57f0290e2048b764de1218470f4f7
Re: Review Request 19731: Patch for KAFKA-1328
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/19731/ --- (Updated April 13, 2014, 1:30 a.m.) Review request for kafka. Bugs: KAFKA-1328 https://issues.apache.org/jira/browse/KAFKA-1328 Repository: kafka Description (updated) --- Changed the signature of poll to return MapString,ConsumerRecordMetadata to organize the ConsumerRecords around topic and then optionally around partition. This will serve the group management as well as custom partition subscription use cases 1. Changed the signature of poll() to return MapString, ListConsumerRecord 2. Changed ConsumerRecord to throw an exception if an error is detected for the partition. For example, if a single large message is larger than the total memory just for that partition, we don't want poll() to throw an exception since that will affect the processing of the remaining partitions as well Fixed MockConsumer to make subscribe(topics) and subscribe(partitions) mutually exclusive Changed the package to org.apache.kafka.clients.consumer from kafka.clients.consumer Changed the package to org.apache.kafka.clients.consumer from kafka.clients.consumer 1. Removed the commitAsync() APIs 2. Changed the commit() APIs to return a Future Fixed configs to match the producer side configs for metrics Renamed AUTO_COMMIT_ENABLE_CONFIG to ENABLE_AUTO_COMMIT_CONFIG Addressing review comments from Tim and Guozhang Rebasing after producer side config cleanup Added license headers Cleaned javadoc for ConsumerConfig Fixed minor indentation in ConsumerConfig Improve docs on ConsumerConfig 1. Added ClientUtils 2. Added basic constructor implementation for KafkaConsumer Improved MockConsumer Chris's feedback and also consumer rewind example code Added commit() and commitAsync() APIs to the consumer and updated docs and examples to reflect that 1. Added consumer usage examples to javadoc 2. Changed signature of APIs that accept or return offsets from list of offsets to map of offsets Improved example for using ConsumerRebalanceCallback Improved example for using ConsumerRebalanceCallback Included Jun's review comments and renamed positions to seek. Also included position() Changes to javadoc for positions() Changed the javadoc for ConsumerRebalanceCallback Changing unsubscribe to also take in var args for topic list Incorporated first round of feedback from Jay, Pradeep and Mattijs on the mailing list Updated configs Javadoc for consumer complete Completed docs for Consumer and ConsumerRebalanceCallback. Added MockConsumer Added the initial interfaces and related documentation for the consumer. More docs required to complete the public API Diffs (updated) - clients/src/main/java/kafka/common/TopicPartitionOffset.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/Consumer.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerConfig.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRebalanceCallback.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRecord.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/KafkaConsumer.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/MockConsumer.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/OffsetMetadata.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/internals/FutureOffsetMetadata.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/producer/KafkaProducer.java a6423f4b37a57f0290e2048b764de1218470f4f7 clients/src/main/java/org/apache/kafka/common/utils/ClientUtils.java PRE-CREATION clients/src/test/java/org/apache/kafka/clients/consumer/ConsumerExampleTest.java PRE-CREATION Diff: https://reviews.apache.org/r/19731/diff/ Testing --- Thanks, Neha Narkhede
Re: Review Request 19731: Patch for KAFKA-1328
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/19731/ --- (Updated April 13, 2014, 2:12 a.m.) Review request for kafka. Bugs: KAFKA-1328 https://issues.apache.org/jira/browse/KAFKA-1328 Repository: kafka Description (updated) --- Fixed the javadoc usage examples in KafkaConsumer to match the API changes Changed the signature of poll to return MapString,ConsumerRecordMetadata to organize the ConsumerRecords around topic and then optionally around partition. This will serve the group management as well as custom partition subscription use cases 1. Changed the signature of poll() to return MapString, ListConsumerRecord 2. Changed ConsumerRecord to throw an exception if an error is detected for the partition. For example, if a single large message is larger than the total memory just for that partition, we don't want poll() to throw an exception since that will affect the processing of the remaining partitions as well Fixed MockConsumer to make subscribe(topics) and subscribe(partitions) mutually exclusive Changed the package to org.apache.kafka.clients.consumer from kafka.clients.consumer Changed the package to org.apache.kafka.clients.consumer from kafka.clients.consumer 1. Removed the commitAsync() APIs 2. Changed the commit() APIs to return a Future Fixed configs to match the producer side configs for metrics Renamed AUTO_COMMIT_ENABLE_CONFIG to ENABLE_AUTO_COMMIT_CONFIG Addressing review comments from Tim and Guozhang Rebasing after producer side config cleanup Added license headers Cleaned javadoc for ConsumerConfig Fixed minor indentation in ConsumerConfig Improve docs on ConsumerConfig 1. Added ClientUtils 2. Added basic constructor implementation for KafkaConsumer Improved MockConsumer Chris's feedback and also consumer rewind example code Added commit() and commitAsync() APIs to the consumer and updated docs and examples to reflect that 1. Added consumer usage examples to javadoc 2. Changed signature of APIs that accept or return offsets from list of offsets to map of offsets Improved example for using ConsumerRebalanceCallback Improved example for using ConsumerRebalanceCallback Included Jun's review comments and renamed positions to seek. Also included position() Changes to javadoc for positions() Changed the javadoc for ConsumerRebalanceCallback Changing unsubscribe to also take in var args for topic list Incorporated first round of feedback from Jay, Pradeep and Mattijs on the mailing list Updated configs Javadoc for consumer complete Completed docs for Consumer and ConsumerRebalanceCallback. Added MockConsumer Added the initial interfaces and related documentation for the consumer. More docs required to complete the public API Diffs (updated) - clients/src/main/java/kafka/common/TopicPartitionOffset.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/Consumer.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerConfig.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRebalanceCallback.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRecord.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/KafkaConsumer.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/MockConsumer.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/OffsetMetadata.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/internals/FutureOffsetMetadata.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/producer/KafkaProducer.java a6423f4b37a57f0290e2048b764de1218470f4f7 clients/src/main/java/org/apache/kafka/common/utils/ClientUtils.java PRE-CREATION clients/src/test/java/org/apache/kafka/clients/consumer/ConsumerExampleTest.java PRE-CREATION Diff: https://reviews.apache.org/r/19731/diff/ Testing --- Thanks, Neha Narkhede
Re: Review Request 19731: Patch for KAFKA-1328
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/19731/ --- (Updated April 11, 2014, 5:54 p.m.) Review request for kafka. Bugs: KAFKA-1328 https://issues.apache.org/jira/browse/KAFKA-1328 Repository: kafka Description (updated) --- Fixed MockConsumer to make subscribe(topics) and subscribe(partitions) mutually exclusive Changed the package to org.apache.kafka.clients.consumer from kafka.clients.consumer Changed the package to org.apache.kafka.clients.consumer from kafka.clients.consumer 1. Removed the commitAsync() APIs 2. Changed the commit() APIs to return a Future Fixed configs to match the producer side configs for metrics Renamed AUTO_COMMIT_ENABLE_CONFIG to ENABLE_AUTO_COMMIT_CONFIG Addressing review comments from Tim and Guozhang Rebasing after producer side config cleanup Added license headers Cleaned javadoc for ConsumerConfig Fixed minor indentation in ConsumerConfig Improve docs on ConsumerConfig 1. Added ClientUtils 2. Added basic constructor implementation for KafkaConsumer Improved MockConsumer Chris's feedback and also consumer rewind example code Added commit() and commitAsync() APIs to the consumer and updated docs and examples to reflect that 1. Added consumer usage examples to javadoc 2. Changed signature of APIs that accept or return offsets from list of offsets to map of offsets Improved example for using ConsumerRebalanceCallback Improved example for using ConsumerRebalanceCallback Included Jun's review comments and renamed positions to seek. Also included position() Changes to javadoc for positions() Changed the javadoc for ConsumerRebalanceCallback Changing unsubscribe to also take in var args for topic list Incorporated first round of feedback from Jay, Pradeep and Mattijs on the mailing list Updated configs Javadoc for consumer complete Completed docs for Consumer and ConsumerRebalanceCallback. Added MockConsumer Added the initial interfaces and related documentation for the consumer. More docs required to complete the public API Diffs (updated) - clients/src/main/java/kafka/common/TopicPartitionOffset.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/Consumer.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerConfig.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRebalanceCallback.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRecord.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/KafkaConsumer.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/MockConsumer.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/OffsetMetadata.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/internals/FutureOffsetMetadata.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/producer/KafkaProducer.java a6423f4b37a57f0290e2048b764de1218470f4f7 clients/src/main/java/org/apache/kafka/common/utils/ClientUtils.java PRE-CREATION clients/src/test/java/org/apache/kafka/clients/consumer/ConsumerExampleTest.java PRE-CREATION Diff: https://reviews.apache.org/r/19731/diff/ Testing --- Thanks, Neha Narkhede
Re: Review Request 19731: Patch for KAFKA-1328
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/19731/ --- (Updated April 11, 2014, 6:16 p.m.) Review request for kafka. Bugs: KAFKA-1328 https://issues.apache.org/jira/browse/KAFKA-1328 Repository: kafka Description (updated) --- 1. Changed the signature of poll() to return MapString, ListConsumerRecord 2. Changed ConsumerRecord to throw an exception if an error is detected for the partition. For example, if a single large message is larger than the total memory just for that partition, we don't want poll() to throw an exception since that will affect the processing of the remaining partitions as well Fixed MockConsumer to make subscribe(topics) and subscribe(partitions) mutually exclusive Changed the package to org.apache.kafka.clients.consumer from kafka.clients.consumer Changed the package to org.apache.kafka.clients.consumer from kafka.clients.consumer 1. Removed the commitAsync() APIs 2. Changed the commit() APIs to return a Future Fixed configs to match the producer side configs for metrics Renamed AUTO_COMMIT_ENABLE_CONFIG to ENABLE_AUTO_COMMIT_CONFIG Addressing review comments from Tim and Guozhang Rebasing after producer side config cleanup Added license headers Cleaned javadoc for ConsumerConfig Fixed minor indentation in ConsumerConfig Improve docs on ConsumerConfig 1. Added ClientUtils 2. Added basic constructor implementation for KafkaConsumer Improved MockConsumer Chris's feedback and also consumer rewind example code Added commit() and commitAsync() APIs to the consumer and updated docs and examples to reflect that 1. Added consumer usage examples to javadoc 2. Changed signature of APIs that accept or return offsets from list of offsets to map of offsets Improved example for using ConsumerRebalanceCallback Improved example for using ConsumerRebalanceCallback Included Jun's review comments and renamed positions to seek. Also included position() Changes to javadoc for positions() Changed the javadoc for ConsumerRebalanceCallback Changing unsubscribe to also take in var args for topic list Incorporated first round of feedback from Jay, Pradeep and Mattijs on the mailing list Updated configs Javadoc for consumer complete Completed docs for Consumer and ConsumerRebalanceCallback. Added MockConsumer Added the initial interfaces and related documentation for the consumer. More docs required to complete the public API Diffs (updated) - clients/src/main/java/kafka/common/TopicPartitionOffset.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/Consumer.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerConfig.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRebalanceCallback.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRecord.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/KafkaConsumer.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/MockConsumer.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/OffsetMetadata.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/internals/FutureOffsetMetadata.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/producer/KafkaProducer.java a6423f4b37a57f0290e2048b764de1218470f4f7 clients/src/main/java/org/apache/kafka/common/utils/ClientUtils.java PRE-CREATION clients/src/test/java/org/apache/kafka/clients/consumer/ConsumerExampleTest.java PRE-CREATION Diff: https://reviews.apache.org/r/19731/diff/ Testing --- Thanks, Neha Narkhede
Re: Review Request 19731: Patch for KAFKA-1328
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/19731/ --- (Updated April 11, 2014, 1:30 a.m.) Review request for kafka. Bugs: KAFKA-1328 https://issues.apache.org/jira/browse/KAFKA-1328 Repository: kafka Description (updated) --- 1. Removed the commitAsync() APIs 2. Changed the commit() APIs to return a Future Fixed configs to match the producer side configs for metrics Renamed AUTO_COMMIT_ENABLE_CONFIG to ENABLE_AUTO_COMMIT_CONFIG Addressing review comments from Tim and Guozhang Rebasing after producer side config cleanup Added license headers Cleaned javadoc for ConsumerConfig Fixed minor indentation in ConsumerConfig Improve docs on ConsumerConfig 1. Added ClientUtils 2. Added basic constructor implementation for KafkaConsumer Improved MockConsumer Chris's feedback and also consumer rewind example code Added commit() and commitAsync() APIs to the consumer and updated docs and examples to reflect that 1. Added consumer usage examples to javadoc 2. Changed signature of APIs that accept or return offsets from list of offsets to map of offsets Improved example for using ConsumerRebalanceCallback Improved example for using ConsumerRebalanceCallback Included Jun's review comments and renamed positions to seek. Also included position() Changes to javadoc for positions() Changed the javadoc for ConsumerRebalanceCallback Changing unsubscribe to also take in var args for topic list Incorporated first round of feedback from Jay, Pradeep and Mattijs on the mailing list Updated configs Javadoc for consumer complete Completed docs for Consumer and ConsumerRebalanceCallback. Added MockConsumer Added the initial interfaces and related documentation for the consumer. More docs required to complete the public API Diffs (updated) - clients/src/main/java/kafka/clients/consumer/Consumer.java PRE-CREATION clients/src/main/java/kafka/clients/consumer/ConsumerConfig.java PRE-CREATION clients/src/main/java/kafka/clients/consumer/ConsumerRebalanceCallback.java PRE-CREATION clients/src/main/java/kafka/clients/consumer/ConsumerRecord.java PRE-CREATION clients/src/main/java/kafka/clients/consumer/KafkaConsumer.java PRE-CREATION clients/src/main/java/kafka/clients/consumer/MockConsumer.java PRE-CREATION clients/src/main/java/kafka/common/TopicPartitionOffset.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/producer/KafkaProducer.java a6423f4b37a57f0290e2048b764de1218470f4f7 clients/src/main/java/org/apache/kafka/common/utils/ClientUtils.java PRE-CREATION clients/src/test/java/org/apache/kafka/clients/consumer/ConsumerExampleTest.java PRE-CREATION Diff: https://reviews.apache.org/r/19731/diff/ Testing --- Thanks, Neha Narkhede
Re: Review Request 19731: Patch for KAFKA-1328
On April 7, 2014, 6:13 p.m., Timothy Chen wrote: clients/src/main/java/kafka/clients/consumer/Consumer.java, line 61 https://reviews.apache.org/r/19731/diff/1/?file=538460#file538460line61 I wonder if it's worth adding more comments about the poll behavior here, about what conditions will it return from poll (ie: when timeout hit, first topic data available, etc). Makes sense. Improved that. On April 7, 2014, 6:13 p.m., Timothy Chen wrote: clients/src/main/java/kafka/clients/consumer/ConsumerConfig.java, line 55 https://reviews.apache.org/r/19731/diff/1/?file=538461#file538461line55 What happens when you explicitly call commit when auto commit is enabled? That is valid. It commits synchronously or asynchronously depending on whether you use commit() or commitAsync(). On April 7, 2014, 6:13 p.m., Timothy Chen wrote: clients/src/main/java/kafka/clients/consumer/ConsumerConfig.java, line 109 https://reviews.apache.org/r/19731/diff/1/?file=538461#file538461line109 Don't we throw an exception instead of being stuck? Being stuck doesn't sounds like a acceptable behavior That's true. This is not ideal. Currently we do not have the ability to stream a single large message and if a message is larger than the largest fetch buffer the consumer can allocate (total.memory.bytes), then the poll() would continue timing out and returning no messages for that partition. So I see that fetch.memory.bytes is not required since we should automatically size the fetch buffer to accommodate large messages. However, if the message is larger than that, there are 2 choices - 1. Throw an exception from poll() indicating a single large message that is larger than (total.memory.bytes) 2. Keep returning no messages from that partition, effectively building up a lag on that partition that the user can only find out about through monitoring. On April 7, 2014, 6:13 p.m., Timothy Chen wrote: clients/src/main/java/kafka/clients/consumer/KafkaConsumer.java, line 113 https://reviews.apache.org/r/19731/diff/1/?file=538464#file538464line113 How does one actually get the list of partitions that returns failure? Here is one way - process() is implemented by the user, so one can keep track of the partitions for which the processing failed and let failedPartitions() return that. On April 7, 2014, 6:13 p.m., Timothy Chen wrote: clients/src/main/java/kafka/clients/consumer/MockConsumer.java, line 51 https://reviews.apache.org/r/19731/diff/1/?file=538465#file538465line51 Looks like the mock consumer doesn't follow the mutually exclusive rule. Good point, this needs to be fixed. On April 7, 2014, 6:13 p.m., Timothy Chen wrote: clients/src/main/java/kafka/clients/consumer/KafkaConsumer.java, line 121 https://reviews.apache.org/r/19731/diff/1/?file=538464#file538464line121 Potentially in this sample code you could throw an exception in your catch block and never call close. Is that ok too? Not ok, but I also don't want to make it excessively complex. I added a try catch and a break to quit the loop - Neha --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/19731/#review39694 --- On March 27, 2014, 4:16 p.m., Neha Narkhede wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/19731/ --- (Updated March 27, 2014, 4:16 p.m.) Review request for kafka. Bugs: KAFKA-1328 https://issues.apache.org/jira/browse/KAFKA-1328 Repository: kafka Description --- Added license headers Cleaned javadoc for ConsumerConfig Fixed minor indentation in ConsumerConfig Improve docs on ConsumerConfig 1. Added ClientUtils 2. Added basic constructor implementation for KafkaConsumer Improved MockConsumer Chris's feedback and also consumer rewind example code Added commit() and commitAsync() APIs to the consumer and updated docs and examples to reflect that 1. Added consumer usage examples to javadoc 2. Changed signature of APIs that accept or return offsets from list of offsets to map of offsets Improved example for using ConsumerRebalanceCallback Improved example for using ConsumerRebalanceCallback Included Jun's review comments and renamed positions to seek. Also included position() Changes to javadoc for positions() Changed the javadoc for ConsumerRebalanceCallback Changing unsubscribe to also take in var args for topic list Incorporated first round of feedback from Jay, Pradeep and Mattijs on the mailing list Updated configs Javadoc for consumer complete Completed docs for Consumer and
Re: Review Request 19731: Patch for KAFKA-1328
On April 7, 2014, 6:03 p.m., Guozhang Wang wrote: clients/src/main/java/kafka/clients/consumer/Consumer.java, line 41 https://reviews.apache.org/r/19731/diff/1/?file=538460#file538460line41 Just wanted to confirm again that we agree to expose TopicPartition to users? Though it is already in common package, it currently only used internally. Right. After writing down some examples using the APIs, I'm convinced that we need to expose TopicPartition. Take a look at the examples to see if you can think of a better solution. On April 7, 2014, 6:03 p.m., Guozhang Wang wrote: clients/src/main/java/kafka/clients/consumer/Consumer.java, line 45 https://reviews.apache.org/r/19731/diff/1/?file=538460#file538460line45 Does this mean subscribe(TopicPartition) followed by an unsubscribe(String) will also throw an error? That's correct. I think it says that it only works in conjunction with subscribe(topics), so yes that will throw some exception. On April 7, 2014, 6:03 p.m., Guozhang Wang wrote: clients/src/main/java/kafka/clients/consumer/Consumer.java, line 54 https://reviews.apache.org/r/19731/diff/1/?file=538460#file538460line54 Ditto as above. Here is what it says - It is an error to unsubscribe from a partition that was never subscribed to using {@link #subscribe(TopicPartition...) subscribe(partitions)} On April 7, 2014, 6:03 p.m., Guozhang Wang wrote: clients/src/main/java/kafka/clients/consumer/Consumer.java, line 77 https://reviews.apache.org/r/19731/diff/1/?file=538460#file538460line77 If no offsets are specified, how does this mean as map values? Or should we say sth like If the specified offset is negative.. That comment was stale given that we also have commit() and commitAsync(). Removed it. On April 7, 2014, 6:03 p.m., Guozhang Wang wrote: clients/src/main/java/kafka/clients/consumer/Consumer.java, line 92 https://reviews.apache.org/r/19731/diff/1/?file=538460#file538460line92 Not sure if we have discussed details about how to implement async commit in the new consumer? It just means the consumer will not wait for the OffsetCommitResponse before returning from commitAsync(). We can discuss details on the wiki, but looks like we have to add this API. I can imagine several applications that would not want to commit synchronously. On April 7, 2014, 6:03 p.m., Guozhang Wang wrote: clients/src/main/java/kafka/clients/consumer/ConsumerConfig.java, line 55 https://reviews.apache.org/r/19731/diff/1/?file=538461#file538461line55 ENABLE_AUTO_COMMIT_CONFIG to be consistent with ENABLE_JMX_CONFIG Well, I removed it and added the corresponding metrics reporter configs to stay consistent with the producer. On April 7, 2014, 6:03 p.m., Guozhang Wang wrote: clients/src/main/java/kafka/clients/consumer/ConsumerConfig.java, line 112 https://reviews.apache.org/r/19731/diff/1/?file=538461#file538461line112 fetch.buffer.bytes I think we might not need this config, but we can revisit when we discuss the memory management On April 7, 2014, 6:03 p.m., Guozhang Wang wrote: clients/src/main/java/kafka/clients/consumer/ConsumerRecord.java, line 22 https://reviews.apache.org/r/19731/diff/1/?file=538463#file538463line22 Suggest add the following functions: compressionType() I'm not so sure that it will make sense. The consumer record is always the original raw record and is always handed out after the decompression anyways. Thoughts? On April 7, 2014, 6:03 p.m., Guozhang Wang wrote: clients/src/main/java/kafka/common/TopicPartitionOffset.java, line 13 https://reviews.apache.org/r/19731/diff/1/?file=538466#file538466line13 This class is currently only used in user customized callback function. So do we really need to provide this class? Nope. After the latest API changes I made, this class can be deleted. On April 7, 2014, 6:03 p.m., Guozhang Wang wrote: clients/src/main/java/kafka/clients/consumer/KafkaConsumer.java, line 337 https://reviews.apache.org/r/19731/diff/1/?file=538464#file538464line337 Two constructors KafkaConsumer(ConsumerConfig config) and KafkaConsumer(ConsumerConfig config, ConsumerRebalanceCallback callback) missing. The latter is there, I added the former On April 7, 2014, 6:03 p.m., Guozhang Wang wrote: clients/src/main/java/kafka/clients/consumer/ConsumerRebalanceCallback.java, lines 26-47 https://reviews.apache.org/r/19731/diff/1/?file=538462#file538462line26 Have we considered the following case: in a poll() function the consumer realized a rebalance is triggered, and hence call onPartitionRevoked and onPartitionAssigned, and then poll() times out, the user then call commit(partitions) on the old partitions. That's a dangerous use of commit(), I think we discussed that the co-ordinator will reject requests to commit offsets for partitions that the consumer id doesn't own. I
Re: Review Request 19731: Patch for KAFKA-1328
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/19731/#review39700 --- clients/src/main/java/kafka/clients/consumer/Consumer.java https://reviews.apache.org/r/19731/#comment72291 subscribe(topic, partitions) = subscribe(partitions) clients/src/main/java/kafka/clients/consumer/Consumer.java https://reviews.apache.org/r/19731/#comment72287 Just wanted to confirm again that we agree to expose TopicPartition to users? Though it is already in common package, it currently only used internally. clients/src/main/java/kafka/clients/consumer/Consumer.java https://reviews.apache.org/r/19731/#comment72288 Does this mean subscribe(TopicPartition) followed by an unsubscribe(String) will also throw an error? clients/src/main/java/kafka/clients/consumer/Consumer.java https://reviews.apache.org/r/19731/#comment72293 Ditto as above. clients/src/main/java/kafka/clients/consumer/Consumer.java https://reviews.apache.org/r/19731/#comment72294 If no offsets are specified, how does this mean as map values? Or should we say sth like If the specified offset is negative.. clients/src/main/java/kafka/clients/consumer/Consumer.java https://reviews.apache.org/r/19731/#comment72295 Ditto above. clients/src/main/java/kafka/clients/consumer/Consumer.java https://reviews.apache.org/r/19731/#comment72296 Not sure if we have discussed details about how to implement async commit in the new consumer? clients/src/main/java/kafka/clients/consumer/Consumer.java https://reviews.apache.org/r/19731/#comment72297 starting fetch offsets, fetch position: inconsistency terms. clients/src/main/java/kafka/clients/consumer/Consumer.java https://reviews.apache.org/r/19731/#comment72298 ...that the consumer currently consumes from = from the API it seems we can get any partitions' offsets? clients/src/main/java/kafka/clients/consumer/Consumer.java https://reviews.apache.org/r/19731/#comment72299 We need to be a bit clearer about the timestamp: it is the timestamp of the message when it reaches the broker, not the consumer. clients/src/main/java/kafka/clients/consumer/ConsumerConfig.java https://reviews.apache.org/r/19731/#comment72300 ENABLE_AUTO_COMMIT_CONFIG to be consistent with ENABLE_JMX_CONFIG clients/src/main/java/kafka/clients/consumer/ConsumerConfig.java https://reviews.apache.org/r/19731/#comment72301 fetch.buffer.bytes clients/src/main/java/kafka/clients/consumer/ConsumerConfig.java https://reviews.apache.org/r/19731/#comment72302 AUTO_OFFSET_RESET_CONFIG is missing. clients/src/main/java/kafka/clients/consumer/ConsumerRebalanceCallback.java https://reviews.apache.org/r/19731/#comment72304 Have we considered the following case: in a poll() function the consumer realized a rebalance is triggered, and hence call onPartitionRevoked and onPartitionAssigned, and then poll() times out, the user then call commit(partitions) on the old partitions. clients/src/main/java/kafka/clients/consumer/ConsumerRecord.java https://reviews.apache.org/r/19731/#comment72306 Suggest add the following functions: compressionType() clients/src/main/java/kafka/clients/consumer/KafkaConsumer.java https://reviews.apache.org/r/19731/#comment72309 Two constructors KafkaConsumer(ConsumerConfig config) and KafkaConsumer(ConsumerConfig config, ConsumerRebalanceCallback callback) missing. clients/src/main/java/kafka/common/TopicPartitionOffset.java https://reviews.apache.org/r/19731/#comment72311 This class is currently only used in user customized callback function. So do we really need to provide this class? - Guozhang Wang On March 27, 2014, 4:16 p.m., Neha Narkhede wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/19731/ --- (Updated March 27, 2014, 4:16 p.m.) Review request for kafka. Bugs: KAFKA-1328 https://issues.apache.org/jira/browse/KAFKA-1328 Repository: kafka Description --- Added license headers Cleaned javadoc for ConsumerConfig Fixed minor indentation in ConsumerConfig Improve docs on ConsumerConfig 1. Added ClientUtils 2. Added basic constructor implementation for KafkaConsumer Improved MockConsumer Chris's feedback and also consumer rewind example code Added commit() and commitAsync() APIs to the consumer and updated docs and examples to reflect that 1. Added consumer usage examples to javadoc 2. Changed signature of APIs that accept or return offsets from list of offsets to map of offsets Improved example for using ConsumerRebalanceCallback Improved example for using ConsumerRebalanceCallback
Re: Review Request 19731: Patch for KAFKA-1328
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/19731/#review39694 --- clients/src/main/java/kafka/clients/consumer/Consumer.java https://reviews.apache.org/r/19731/#comment72276 I wonder if it's worth adding more comments about the poll behavior here, about what conditions will it return from poll (ie: when timeout hit, first topic data available, etc). clients/src/main/java/kafka/clients/consumer/Consumer.java https://reviews.apache.org/r/19731/#comment72303 I know we return custom future objects for the producer, is there a reason we don't keep it consistent on the consumer? clients/src/main/java/kafka/clients/consumer/ConsumerConfig.java https://reviews.apache.org/r/19731/#comment72277 What happens when you explicitly call commit when auto commit is enabled? clients/src/main/java/kafka/clients/consumer/ConsumerConfig.java https://reviews.apache.org/r/19731/#comment72278 Don't we throw an exception instead of being stuck? Being stuck doesn't sounds like a acceptable behavior clients/src/main/java/kafka/clients/consumer/KafkaConsumer.java https://reviews.apache.org/r/19731/#comment72305 How does one actually get the list of partitions that returns failure? clients/src/main/java/kafka/clients/consumer/KafkaConsumer.java https://reviews.apache.org/r/19731/#comment72279 Potentially in this sample code you could throw an exception in your catch block and never call close. Is that ok too? clients/src/main/java/kafka/clients/consumer/MockConsumer.java https://reviews.apache.org/r/19731/#comment72313 Looks like the mock consumer doesn't follow the mutually exclusive rule. - Timothy Chen On March 27, 2014, 4:16 p.m., Neha Narkhede wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/19731/ --- (Updated March 27, 2014, 4:16 p.m.) Review request for kafka. Bugs: KAFKA-1328 https://issues.apache.org/jira/browse/KAFKA-1328 Repository: kafka Description --- Added license headers Cleaned javadoc for ConsumerConfig Fixed minor indentation in ConsumerConfig Improve docs on ConsumerConfig 1. Added ClientUtils 2. Added basic constructor implementation for KafkaConsumer Improved MockConsumer Chris's feedback and also consumer rewind example code Added commit() and commitAsync() APIs to the consumer and updated docs and examples to reflect that 1. Added consumer usage examples to javadoc 2. Changed signature of APIs that accept or return offsets from list of offsets to map of offsets Improved example for using ConsumerRebalanceCallback Improved example for using ConsumerRebalanceCallback Included Jun's review comments and renamed positions to seek. Also included position() Changes to javadoc for positions() Changed the javadoc for ConsumerRebalanceCallback Changing unsubscribe to also take in var args for topic list Incorporated first round of feedback from Jay, Pradeep and Mattijs on the mailing list Updated configs Javadoc for consumer complete Completed docs for Consumer and ConsumerRebalanceCallback. Added MockConsumer Added the initial interfaces and related documentation for the consumer. More docs required to complete the public API Diffs - clients/src/main/java/kafka/clients/consumer/Consumer.java PRE-CREATION clients/src/main/java/kafka/clients/consumer/ConsumerConfig.java PRE-CREATION clients/src/main/java/kafka/clients/consumer/ConsumerRebalanceCallback.java PRE-CREATION clients/src/main/java/kafka/clients/consumer/ConsumerRecord.java PRE-CREATION clients/src/main/java/kafka/clients/consumer/KafkaConsumer.java PRE-CREATION clients/src/main/java/kafka/clients/consumer/MockConsumer.java PRE-CREATION clients/src/main/java/kafka/common/TopicPartitionOffset.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/producer/KafkaProducer.java 1ff9174870a8c9cd97eb6655416edd4124377b0e clients/src/main/java/org/apache/kafka/common/utils/ClientUtils.java PRE-CREATION clients/src/test/java/org/apache/kafka/clients/consumer/ConsumerExampleTest.java PRE-CREATION Diff: https://reviews.apache.org/r/19731/diff/ Testing --- Thanks, Neha Narkhede
Review Request 19731: Patch for KAFKA-1328
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/19731/ --- Review request for kafka. Bugs: KAFKA-1328 https://issues.apache.org/jira/browse/KAFKA-1328 Repository: kafka Description --- Added license headers Cleaned javadoc for ConsumerConfig Fixed minor indentation in ConsumerConfig Improve docs on ConsumerConfig 1. Added ClientUtils 2. Added basic constructor implementation for KafkaConsumer Improved MockConsumer Chris's feedback and also consumer rewind example code Added commit() and commitAsync() APIs to the consumer and updated docs and examples to reflect that 1. Added consumer usage examples to javadoc 2. Changed signature of APIs that accept or return offsets from list of offsets to map of offsets Improved example for using ConsumerRebalanceCallback Improved example for using ConsumerRebalanceCallback Included Jun's review comments and renamed positions to seek. Also included position() Changes to javadoc for positions() Changed the javadoc for ConsumerRebalanceCallback Changing unsubscribe to also take in var args for topic list Incorporated first round of feedback from Jay, Pradeep and Mattijs on the mailing list Updated configs Javadoc for consumer complete Completed docs for Consumer and ConsumerRebalanceCallback. Added MockConsumer Added the initial interfaces and related documentation for the consumer. More docs required to complete the public API Diffs - clients/src/main/java/kafka/clients/consumer/Consumer.java PRE-CREATION clients/src/main/java/kafka/clients/consumer/ConsumerConfig.java PRE-CREATION clients/src/main/java/kafka/clients/consumer/ConsumerRebalanceCallback.java PRE-CREATION clients/src/main/java/kafka/clients/consumer/ConsumerRecord.java PRE-CREATION clients/src/main/java/kafka/clients/consumer/KafkaConsumer.java PRE-CREATION clients/src/main/java/kafka/clients/consumer/MockConsumer.java PRE-CREATION clients/src/main/java/kafka/common/TopicPartitionOffset.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/producer/KafkaProducer.java 1ff9174870a8c9cd97eb6655416edd4124377b0e clients/src/main/java/org/apache/kafka/common/utils/ClientUtils.java PRE-CREATION clients/src/test/java/org/apache/kafka/clients/consumer/ConsumerExampleTest.java PRE-CREATION Diff: https://reviews.apache.org/r/19731/diff/ Testing --- Thanks, Neha Narkhede