Re: Review Request 20997: Fix KAFKA-1432
On May 2, 2014, 1:35 a.m., Jun Rao wrote: perf/src/main/scala/kafka/perf/ProducerPerformance.scala, lines 308-309 https://reviews.apache.org/r/20997/diff/1/?file=573270#file573270line308 Is there a reason that we change to send a null key? Will that break system tests? In the original interface that value is actually used as the partition id instead of the key, and in the old producer it is hashed in the DefaultEventHandler, while for the new producer it is hashed before calling the send() call. With the null key passed in, the old producer will have sticky behavior while the new producer's round robin stays the same. I will convert the integer to byte array to make the producer also round robin. - Guozhang --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/20997/#review41998 --- On May 1, 2014, 11:46 p.m., Guozhang Wang wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/20997/ --- (Updated May 1, 2014, 11:46 p.m.) Review request for kafka. Bugs: KAFKA-1432 https://issues.apache.org/jira/browse/KAFKA-1432 Repository: kafka Description --- 1. Add useNewProducer in the MirrorMaker. 2. Remove MirrorMaker's dependency on MigrationTool's producer thread and data channel. 3. Unify Old/New Producer in a new class for MirrorMaker, ProducerPerformance and ConsoleProducer Will remove kafka.tools.newproducer.MirrorMaker upon final commit. Diffs - core/src/main/scala/kafka/producer/ByteArrayPartitioner.scala 988e4374d8c9b30c969741c30c75f0c44f2c core/src/main/scala/kafka/producer/ConsoleProducer.scala b19ab49c5fd57423e142f2f8afc5a77e653fd6ed core/src/main/scala/kafka/producer/TransitProducer.scala PRE-CREATION core/src/main/scala/kafka/tools/MirrorMaker.scala e4d1a86f673f7df7fe71ce44e3550fcde8d01bba perf/src/main/scala/kafka/perf/ProducerPerformance.scala 1490bdb3d52d5c901d7a46484935bda2f7ca Diff: https://reviews.apache.org/r/20997/diff/ Testing --- Thanks, Guozhang Wang
Re: Review Request 20997: Fix KAFKA-1432
On May 2, 2014, 5:51 p.m., Neha Narkhede wrote: core/src/main/scala/kafka/producer/TransitProducer.scala, lines 24-25 https://reviews.apache.org/r/20997/diff/1/?file=573268#file573268line24 +1 on renaming this as per Jun's suggestion. If this is only for the tools, then let's move it to the tools package. ConsoleProducer is in kafka.producer ProducerPerformance is in kafka.perf MirrorMaker is in kafka.tools So I decided to put the BaseProducer in kafka.producer. Let me know if you have another preference. On May 2, 2014, 5:51 p.m., Neha Narkhede wrote: core/src/main/scala/kafka/producer/ConsoleProducer.scala, line 50 https://reviews.apache.org/r/20997/diff/1/?file=573267#file573267line50 It will be cleaner to move this logic into a buildProducerConfig private API that builds either the new or old producer's configs based on some boolean value. The thing is that this buildProducerConfig would not be able to be shared between ConsoleProducer and ProducerPerformance since the former used its own ProducerConfig while the latter used its own ProducerPerfConfig. So we would have two buildProducerConfig functions in these two classes and just move the logic there. I think it would not help much for the cleaning. - Guozhang --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/20997/#review42028 --- On May 1, 2014, 11:46 p.m., Guozhang Wang wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/20997/ --- (Updated May 1, 2014, 11:46 p.m.) Review request for kafka. Bugs: KAFKA-1432 https://issues.apache.org/jira/browse/KAFKA-1432 Repository: kafka Description --- 1. Add useNewProducer in the MirrorMaker. 2. Remove MirrorMaker's dependency on MigrationTool's producer thread and data channel. 3. Unify Old/New Producer in a new class for MirrorMaker, ProducerPerformance and ConsoleProducer Will remove kafka.tools.newproducer.MirrorMaker upon final commit. Diffs - core/src/main/scala/kafka/producer/ByteArrayPartitioner.scala 988e4374d8c9b30c969741c30c75f0c44f2c core/src/main/scala/kafka/producer/ConsoleProducer.scala b19ab49c5fd57423e142f2f8afc5a77e653fd6ed core/src/main/scala/kafka/producer/TransitProducer.scala PRE-CREATION core/src/main/scala/kafka/tools/MirrorMaker.scala e4d1a86f673f7df7fe71ce44e3550fcde8d01bba perf/src/main/scala/kafka/perf/ProducerPerformance.scala 1490bdb3d52d5c901d7a46484935bda2f7ca Diff: https://reviews.apache.org/r/20997/diff/ Testing --- Thanks, Guozhang Wang
Re: Review Request 20745: Patch for KAFKA-1397
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/20745/#review42058 --- core/src/main/scala/kafka/controller/TopicDeletionManager.scala https://reviews.apache.org/r/20745/#comment75875 Perhaps it's better to move them to after deleteTopicsThread.awaitShutdown() since only then we are certain that the delete topics thread is gone. core/src/main/scala/kafka/controller/TopicDeletionManager.scala https://reviews.apache.org/r/20745/#comment75876 Perhaps this can be renamed as isPartitionToBeDeleted. core/src/main/scala/kafka/controller/TopicDeletionManager.scala https://reviews.apache.org/r/20745/#comment75877 This should be debug level. core/src/main/scala/kafka/controller/TopicDeletionManager.scala https://reviews.apache.org/r/20745/#comment75882 This comment is no longer valid. core/src/main/scala/kafka/utils/ShutdownableThread.scala https://reviews.apache.org/r/20745/#comment75878 I thought you plan to remove isShuttingDown? core/src/main/scala/kafka/log/LogManager.scala https://reviews.apache.org/r/20745/#comment75798 You can revert the changes in LogManager now. - Jun Rao On May 2, 2014, 8:38 p.m., Timothy Chen wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/20745/ --- (Updated May 2, 2014, 8:38 p.m.) Review request for kafka. Bugs: KAFKA-1397 https://issues.apache.org/jira/browse/KAFKA-1397 Repository: kafka Description --- KAFKA-1397: Fix delete topic tests and deadlock Diffs - core/src/main/scala/kafka/admin/AdminUtils.scala 36ddeb44490e8343a4e8056c45726ac660e4b2f9 core/src/main/scala/kafka/controller/ControllerChannelManager.scala 919aeb26f93d2fc34d873cfb3dbfab7235a9d635 core/src/main/scala/kafka/controller/KafkaController.scala 933de9dd324c7086efe6aa610335ef370d9e9c12 core/src/main/scala/kafka/controller/ReplicaStateMachine.scala 0e47dac8cbf65a86d053a3371a18af467afd70ae core/src/main/scala/kafka/controller/TopicDeletionManager.scala e4bc2439ce1933c7c7571d255464ee678226a6cb core/src/main/scala/kafka/log/LogManager.scala ac67f081e6219fd2181479e7a2bb88ea6044e6cc core/src/main/scala/kafka/server/ReplicaManager.scala 11c20cee83fda9a492156674d351a0096b13fd99 core/src/main/scala/kafka/utils/ShutdownableThread.scala cf8adc9f468f4d6f01d1303efe39a3ec6f3d9b53 core/src/test/scala/unit/kafka/admin/DeleteTopicTest.scala 9c29e144bba2c9bafa91941b6ca5c263490693b3 core/src/test/scala/unit/kafka/server/DynamicConfigChangeTest.scala 5c487968014b56490eb2bc876cef1c52efd1cdad core/src/test/scala/unit/kafka/server/ServerShutdownTest.scala 014e9644dba1d65142f6a9abc745858870a46230 core/src/test/scala/unit/kafka/utils/TestUtils.scala 00bfba41fe008fe72c17fb788e3cada3daf8eb30 Diff: https://reviews.apache.org/r/20745/diff/ Testing --- Thanks, Timothy Chen
Re: Review Request 20997: Fix KAFKA-1432: address Jun's comments
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/20997/ --- (Updated May 5, 2014, 5:31 p.m.) Review request for kafka. Summary (updated) - Fix KAFKA-1432: address Jun's comments Bugs: KAFKA-1432 https://issues.apache.org/jira/browse/KAFKA-1432 Repository: kafka Description (updated) --- Address Jun's comments Add new TransitProducer KAFKA-1432.v1 Diffs (updated) - core/src/main/scala/kafka/producer/BaseProducer.scala PRE-CREATION core/src/main/scala/kafka/producer/ByteArrayPartitioner.scala 988e4374d8c9b30c969741c30c75f0c44f2c core/src/main/scala/kafka/producer/ConsoleProducer.scala b19ab49c5fd57423e142f2f8afc5a77e653fd6ed core/src/main/scala/kafka/tools/MirrorMaker.scala e4d1a86f673f7df7fe71ce44e3550fcde8d01bba perf/src/main/scala/kafka/perf/ProducerPerformance.scala 1490bdb3d52d5c901d7a46484935bda2f7ca system_test/utils/kafka_system_test_utils.py 423b512983c77b41b819912fe67b375a78faf644 Diff: https://reviews.apache.org/r/20997/diff/ Testing --- Thanks, Guozhang Wang
[jira] [Updated] (KAFKA-1432) Make num.producerThreads configurable on new MirrrorMaker
[ https://issues.apache.org/jira/browse/KAFKA-1432?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Guozhang Wang updated KAFKA-1432: - Attachment: KAFKA-1432_2014-05-05_10:31:42.patch Make num.producerThreads configurable on new MirrrorMaker - Key: KAFKA-1432 URL: https://issues.apache.org/jira/browse/KAFKA-1432 Project: Kafka Issue Type: Bug Reporter: Guozhang Wang Assignee: Guozhang Wang Fix For: 0.8.2 Attachments: KAFKA-1432.patch, KAFKA-1432_2014-05-05_10:31:42.patch Originally we make the num.producerThreads coupled with num.consumerThreads since we thought with the new producer, a single or couple of instances are sufficient for throughput. However, with compression turned on, which is now executed in the caller thread we still need configurable number of producer threads. -- This message was sent by Atlassian JIRA (v6.2#6252)
Re: Review Request 20718: Patch for KAFKA-1384
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/20718/#review42169 --- core/src/main/scala/kafka/server/BrokerStates.scala https://reviews.apache.org/r/20718/#comment75904 This diagram is very helpful. We need an arrow from RunningAsController to RendingControlledShutdown. - Jun Rao On May 2, 2014, 7:13 p.m., Timothy Chen wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/20718/ --- (Updated May 2, 2014, 7:13 p.m.) Review request for kafka. Bugs: KAFKA-1384 https://issues.apache.org/jira/browse/KAFKA-1384 Repository: kafka Description --- KAFKA-1384: Logging kafka state metric Diffs - core/src/main/scala/kafka/controller/KafkaController.scala 933de9dd324c7086efe6aa610335ef370d9e9c12 core/src/main/scala/kafka/log/Log.scala b7bc5ffdecba7221b3d2e4bca2b0c703bc74fa12 core/src/main/scala/kafka/log/LogManager.scala ac67f081e6219fd2181479e7a2bb88ea6044e6cc core/src/main/scala/kafka/server/BrokerStates.scala PRE-CREATION core/src/main/scala/kafka/server/KafkaServer.scala c208f83bed7fb91f07fae42f2b66892e6d46fecc core/src/main/scala/kafka/server/KafkaServerStartable.scala acda52b801714bcc182edc0ced925f0e4b493fc1 Diff: https://reviews.apache.org/r/20718/diff/ Testing --- Thanks, Timothy Chen
[jira] [Commented] (KAFKA-1432) Make num.producerThreads configurable on new MirrrorMaker
[ https://issues.apache.org/jira/browse/KAFKA-1432?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=13989743#comment-13989743 ] Guozhang Wang commented on KAFKA-1432: -- Updated reviewboard https://reviews.apache.org/r/20997/ against branch origin/trunk Make num.producerThreads configurable on new MirrrorMaker - Key: KAFKA-1432 URL: https://issues.apache.org/jira/browse/KAFKA-1432 Project: Kafka Issue Type: Bug Reporter: Guozhang Wang Assignee: Guozhang Wang Fix For: 0.8.2 Attachments: KAFKA-1432.patch, KAFKA-1432_2014-05-05_10:31:42.patch Originally we make the num.producerThreads coupled with num.consumerThreads since we thought with the new producer, a single or couple of instances are sufficient for throughput. However, with compression turned on, which is now executed in the caller thread we still need configurable number of producer threads. -- This message was sent by Atlassian JIRA (v6.2#6252)
Re: Review Request 20718: Patch for KAFKA-1384
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/20718/ --- (Updated May 5, 2014, 6:05 p.m.) Review request for kafka. Bugs: KAFKA-1384 https://issues.apache.org/jira/browse/KAFKA-1384 Repository: kafka Description --- KAFKA-1384: Logging kafka state metric Diffs (updated) - core/src/main/scala/kafka/controller/KafkaController.scala 933de9dd324c7086efe6aa610335ef370d9e9c12 core/src/main/scala/kafka/log/Log.scala b7bc5ffdecba7221b3d2e4bca2b0c703bc74fa12 core/src/main/scala/kafka/log/LogManager.scala ac67f081e6219fd2181479e7a2bb88ea6044e6cc core/src/main/scala/kafka/server/BrokerStates.scala PRE-CREATION core/src/main/scala/kafka/server/KafkaServer.scala c208f83bed7fb91f07fae42f2b66892e6d46fecc core/src/main/scala/kafka/server/KafkaServerStartable.scala acda52b801714bcc182edc0ced925f0e4b493fc1 Diff: https://reviews.apache.org/r/20718/diff/ Testing --- Thanks, Timothy Chen
[jira] [Commented] (KAFKA-1384) Log Broker state
[ https://issues.apache.org/jira/browse/KAFKA-1384?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=13989785#comment-13989785 ] Timothy Chen commented on KAFKA-1384: - Updated reviewboard https://reviews.apache.org/r/20718/ against branch origin/trunk Log Broker state - Key: KAFKA-1384 URL: https://issues.apache.org/jira/browse/KAFKA-1384 Project: Kafka Issue Type: New Feature Reporter: Timothy Chen Assignee: Timothy Chen Attachments: KAFKA-1384.patch, KAFKA-1384_2014-04-26_00:09:56.patch, KAFKA-1384_2014-05-01_18:47:03.patch, KAFKA-1384_2014-05-02_12:13:24.patch, KAFKA-1384_2014-05-05_11:05:15.patch Currently we don't have visibility into what state the broker is currently in, ie: Startup - Running - Waiting Controlled shutdown - Shutting down So without knowing what state the broker it is it's hard to figure out what the current broker is performing. This ticket is to add a new metric to expose the current broker state. -- This message was sent by Atlassian JIRA (v6.2#6252)
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 20745: Patch for KAFKA-1397
On May 5, 2014, 5:12 p.m., Jun Rao wrote: core/src/main/scala/kafka/utils/ShutdownableThread.scala, lines 28-33 https://reviews.apache.org/r/20745/diff/6/?file=573313#file573313line28 I thought you plan to remove isShuttingDown? Odd I don't have this anymore in my box, but end up in my patch - Timothy --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/20745/#review42058 --- On May 2, 2014, 8:38 p.m., Timothy Chen wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/20745/ --- (Updated May 2, 2014, 8:38 p.m.) Review request for kafka. Bugs: KAFKA-1397 https://issues.apache.org/jira/browse/KAFKA-1397 Repository: kafka Description --- KAFKA-1397: Fix delete topic tests and deadlock Diffs - core/src/main/scala/kafka/admin/AdminUtils.scala 36ddeb44490e8343a4e8056c45726ac660e4b2f9 core/src/main/scala/kafka/controller/ControllerChannelManager.scala 919aeb26f93d2fc34d873cfb3dbfab7235a9d635 core/src/main/scala/kafka/controller/KafkaController.scala 933de9dd324c7086efe6aa610335ef370d9e9c12 core/src/main/scala/kafka/controller/ReplicaStateMachine.scala 0e47dac8cbf65a86d053a3371a18af467afd70ae core/src/main/scala/kafka/controller/TopicDeletionManager.scala e4bc2439ce1933c7c7571d255464ee678226a6cb core/src/main/scala/kafka/log/LogManager.scala ac67f081e6219fd2181479e7a2bb88ea6044e6cc core/src/main/scala/kafka/server/ReplicaManager.scala 11c20cee83fda9a492156674d351a0096b13fd99 core/src/main/scala/kafka/utils/ShutdownableThread.scala cf8adc9f468f4d6f01d1303efe39a3ec6f3d9b53 core/src/test/scala/unit/kafka/admin/DeleteTopicTest.scala 9c29e144bba2c9bafa91941b6ca5c263490693b3 core/src/test/scala/unit/kafka/server/DynamicConfigChangeTest.scala 5c487968014b56490eb2bc876cef1c52efd1cdad core/src/test/scala/unit/kafka/server/ServerShutdownTest.scala 014e9644dba1d65142f6a9abc745858870a46230 core/src/test/scala/unit/kafka/utils/TestUtils.scala 00bfba41fe008fe72c17fb788e3cada3daf8eb30 Diff: https://reviews.apache.org/r/20745/diff/ Testing --- Thanks, Timothy Chen
[jira] [Commented] (KAFKA-1397) delete topic is not working
[ https://issues.apache.org/jira/browse/KAFKA-1397?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=13989793#comment-13989793 ] Timothy Chen commented on KAFKA-1397: - Updated reviewboard https://reviews.apache.org/r/20745/ against branch origin/trunk delete topic is not working Key: KAFKA-1397 URL: https://issues.apache.org/jira/browse/KAFKA-1397 Project: Kafka Issue Type: Bug Affects Versions: 0.8.2 Reporter: Jun Rao Attachments: KAFKA-1397.patch, KAFKA-1397_2014-04-28_14:48:32.patch, KAFKA-1397_2014-04-28_17:08:49.patch, KAFKA-1397_2014-04-30_14:55:28.patch, KAFKA-1397_2014-05-01_15:53:57.patch, KAFKA-1397_2014-05-01_18:12:24.patch, KAFKA-1397_2014-05-02_13:38:02.patch, KAFKA-1397_2014-05-05_11:17:59.patch All unit tests are disabled since they hang transiently (see details in KAFKA-1391). -- This message was sent by Atlassian JIRA (v6.2#6252)
[jira] [Updated] (KAFKA-1397) delete topic is not working
[ https://issues.apache.org/jira/browse/KAFKA-1397?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Timothy Chen updated KAFKA-1397: Attachment: KAFKA-1397_2014-05-05_11:17:59.patch delete topic is not working Key: KAFKA-1397 URL: https://issues.apache.org/jira/browse/KAFKA-1397 Project: Kafka Issue Type: Bug Affects Versions: 0.8.2 Reporter: Jun Rao Attachments: KAFKA-1397.patch, KAFKA-1397_2014-04-28_14:48:32.patch, KAFKA-1397_2014-04-28_17:08:49.patch, KAFKA-1397_2014-04-30_14:55:28.patch, KAFKA-1397_2014-05-01_15:53:57.patch, KAFKA-1397_2014-05-01_18:12:24.patch, KAFKA-1397_2014-05-02_13:38:02.patch, KAFKA-1397_2014-05-05_11:17:59.patch All unit tests are disabled since they hang transiently (see details in KAFKA-1391). -- This message was sent by Atlassian JIRA (v6.2#6252)
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
[jira] [Commented] (KAFKA-1328) Add new consumer APIs
[ https://issues.apache.org/jira/browse/KAFKA-1328?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=13989817#comment-13989817 ] Neha Narkhede commented on KAFKA-1328: -- Updated reviewboard https://reviews.apache.org/r/19731/ against branch trunk Add new consumer APIs - Key: KAFKA-1328 URL: https://issues.apache.org/jira/browse/KAFKA-1328 Project: Kafka Issue Type: Sub-task Components: consumer Affects Versions: 0.9.0 Reporter: Neha Narkhede Assignee: Neha Narkhede Attachments: KAFKA-1328.patch, KAFKA-1328_2014-04-10_17:13:24.patch, KAFKA-1328_2014-04-10_18:30:48.patch, KAFKA-1328_2014-04-11_10:54:19.patch, KAFKA-1328_2014-04-11_11:16:44.patch, KAFKA-1328_2014-04-12_18:30:22.patch, KAFKA-1328_2014-04-12_19:12:12.patch, KAFKA-1328_2014-05-05_11:35:07.patch New consumer API discussion is here - http://mail-archives.apache.org/mod_mbox/kafka-users/201402.mbox/%3CCAOG_4QYBHwyi0xN=hl1fpnrtkvfjzx14ujfntft3nn_mw3+...@mail.gmail.com%3E This JIRA includes reviewing and checking in the new consumer APIs -- This message was sent by Atlassian JIRA (v6.2#6252)
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
[jira] [Updated] (KAFKA-1328) Add new consumer APIs
[ https://issues.apache.org/jira/browse/KAFKA-1328?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Neha Narkhede updated KAFKA-1328: - Attachment: KAFKA-1328_2014-05-05_11:35:41.patch Add new consumer APIs - Key: KAFKA-1328 URL: https://issues.apache.org/jira/browse/KAFKA-1328 Project: Kafka Issue Type: Sub-task Components: consumer Affects Versions: 0.9.0 Reporter: Neha Narkhede Assignee: Neha Narkhede Attachments: KAFKA-1328.patch, KAFKA-1328_2014-04-10_17:13:24.patch, KAFKA-1328_2014-04-10_18:30:48.patch, KAFKA-1328_2014-04-11_10:54:19.patch, KAFKA-1328_2014-04-11_11:16:44.patch, KAFKA-1328_2014-04-12_18:30:22.patch, KAFKA-1328_2014-04-12_19:12:12.patch, KAFKA-1328_2014-05-05_11:35:07.patch, KAFKA-1328_2014-05-05_11:35:41.patch New consumer API discussion is here - http://mail-archives.apache.org/mod_mbox/kafka-users/201402.mbox/%3CCAOG_4QYBHwyi0xN=hl1fpnrtkvfjzx14ujfntft3nn_mw3+...@mail.gmail.com%3E This JIRA includes reviewing and checking in the new consumer APIs -- This message was sent by Atlassian JIRA (v6.2#6252)
[jira] [Commented] (KAFKA-1328) Add new consumer APIs
[ https://issues.apache.org/jira/browse/KAFKA-1328?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=13989818#comment-13989818 ] Neha Narkhede commented on KAFKA-1328: -- Updated reviewboard https://reviews.apache.org/r/19731/ against branch trunk Add new consumer APIs - Key: KAFKA-1328 URL: https://issues.apache.org/jira/browse/KAFKA-1328 Project: Kafka Issue Type: Sub-task Components: consumer Affects Versions: 0.9.0 Reporter: Neha Narkhede Assignee: Neha Narkhede Attachments: KAFKA-1328.patch, KAFKA-1328_2014-04-10_17:13:24.patch, KAFKA-1328_2014-04-10_18:30:48.patch, KAFKA-1328_2014-04-11_10:54:19.patch, KAFKA-1328_2014-04-11_11:16:44.patch, KAFKA-1328_2014-04-12_18:30:22.patch, KAFKA-1328_2014-04-12_19:12:12.patch, KAFKA-1328_2014-05-05_11:35:07.patch, KAFKA-1328_2014-05-05_11:35:41.patch New consumer API discussion is here - http://mail-archives.apache.org/mod_mbox/kafka-users/201402.mbox/%3CCAOG_4QYBHwyi0xN=hl1fpnrtkvfjzx14ujfntft3nn_mw3+...@mail.gmail.com%3E This JIRA includes reviewing and checking in the new consumer APIs -- This message was sent by Atlassian JIRA (v6.2#6252)
[jira] [Updated] (KAFKA-1328) Add new consumer APIs
[ https://issues.apache.org/jira/browse/KAFKA-1328?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Neha Narkhede updated KAFKA-1328: - Attachment: KAFKA-1328_2014-05-05_11:35:07.patch Add new consumer APIs - Key: KAFKA-1328 URL: https://issues.apache.org/jira/browse/KAFKA-1328 Project: Kafka Issue Type: Sub-task Components: consumer Affects Versions: 0.9.0 Reporter: Neha Narkhede Assignee: Neha Narkhede Attachments: KAFKA-1328.patch, KAFKA-1328_2014-04-10_17:13:24.patch, KAFKA-1328_2014-04-10_18:30:48.patch, KAFKA-1328_2014-04-11_10:54:19.patch, KAFKA-1328_2014-04-11_11:16:44.patch, KAFKA-1328_2014-04-12_18:30:22.patch, KAFKA-1328_2014-04-12_19:12:12.patch, KAFKA-1328_2014-05-05_11:35:07.patch New consumer API discussion is here - http://mail-archives.apache.org/mod_mbox/kafka-users/201402.mbox/%3CCAOG_4QYBHwyi0xN=hl1fpnrtkvfjzx14ujfntft3nn_mw3+...@mail.gmail.com%3E This JIRA includes reviewing and checking in the new consumer APIs -- This message was sent by Atlassian JIRA (v6.2#6252)
Re: Review Request 20997: Fix KAFKA-1432: address Jun's comments
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/20997/#review42185 --- core/src/main/scala/kafka/producer/BaseProducer.scala https://reviews.apache.org/r/20997/#comment75926 transit - base core/src/main/scala/kafka/producer/BaseProducer.scala https://reviews.apache.org/r/20997/#comment75927 sync send - send synchronously core/src/main/scala/kafka/producer/BaseProducer.scala https://reviews.apache.org/r/20997/#comment75928 The comment says block on a full buffer, but the config turns it off. core/src/main/scala/kafka/tools/MirrorMaker.scala https://reviews.apache.org/r/20997/#comment75932 Now, there is a discrepancy in the names. MirrorMakerThread is too general and it is being referred to as consumerThread elsewhere. Should we just rename MirrorMakerThread to ConsumerThread? core/src/main/scala/kafka/tools/MirrorMaker.scala https://reviews.apache.org/r/20997/#comment75930 i-producerIndex - Neha Narkhede On May 5, 2014, 5:31 p.m., Guozhang Wang wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/20997/ --- (Updated May 5, 2014, 5:31 p.m.) Review request for kafka. Bugs: KAFKA-1432 https://issues.apache.org/jira/browse/KAFKA-1432 Repository: kafka Description --- Address Jun's comments Add new TransitProducer KAFKA-1432.v1 Diffs - core/src/main/scala/kafka/producer/BaseProducer.scala PRE-CREATION core/src/main/scala/kafka/producer/ByteArrayPartitioner.scala 988e4374d8c9b30c969741c30c75f0c44f2c core/src/main/scala/kafka/producer/ConsoleProducer.scala b19ab49c5fd57423e142f2f8afc5a77e653fd6ed core/src/main/scala/kafka/tools/MirrorMaker.scala e4d1a86f673f7df7fe71ce44e3550fcde8d01bba perf/src/main/scala/kafka/perf/ProducerPerformance.scala 1490bdb3d52d5c901d7a46484935bda2f7ca system_test/utils/kafka_system_test_utils.py 423b512983c77b41b819912fe67b375a78faf644 Diff: https://reviews.apache.org/r/20997/diff/ Testing --- Thanks, Guozhang Wang
Re: Review Request 20997: Fix KAFKA-1432: address Jun's comments
On May 2, 2014, 5:51 p.m., Neha Narkhede wrote: core/src/main/scala/kafka/producer/ConsoleProducer.scala, line 50 https://reviews.apache.org/r/20997/diff/1/?file=573267#file573267line50 It will be cleaner to move this logic into a buildProducerConfig private API that builds either the new or old producer's configs based on some boolean value. Guozhang Wang wrote: The thing is that this buildProducerConfig would not be able to be shared between ConsoleProducer and ProducerPerformance since the former used its own ProducerConfig while the latter used its own ProducerPerfConfig. So we would have two buildProducerConfig functions in these two classes and just move the logic there. I think it would not help much for the cleaning. makes sense. On May 2, 2014, 5:51 p.m., Neha Narkhede wrote: core/src/main/scala/kafka/producer/TransitProducer.scala, lines 24-25 https://reviews.apache.org/r/20997/diff/1/?file=573268#file573268line24 +1 on renaming this as per Jun's suggestion. If this is only for the tools, then let's move it to the tools package. Guozhang Wang wrote: ConsoleProducer is in kafka.producer ProducerPerformance is in kafka.perf MirrorMaker is in kafka.tools So I decided to put the BaseProducer in kafka.producer. Let me know if you have another preference. ConsoleProducer is a tool and should be moved to kafka.tools ProducerPerformance is also supposed to be in kafka.tools. It seems to me that it's better to do these moves and also move BaseProducer to kafka.tools. - Neha --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/20997/#review42028 --- On May 5, 2014, 5:31 p.m., Guozhang Wang wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/20997/ --- (Updated May 5, 2014, 5:31 p.m.) Review request for kafka. Bugs: KAFKA-1432 https://issues.apache.org/jira/browse/KAFKA-1432 Repository: kafka Description --- Address Jun's comments Add new TransitProducer KAFKA-1432.v1 Diffs - core/src/main/scala/kafka/producer/BaseProducer.scala PRE-CREATION core/src/main/scala/kafka/producer/ByteArrayPartitioner.scala 988e4374d8c9b30c969741c30c75f0c44f2c core/src/main/scala/kafka/producer/ConsoleProducer.scala b19ab49c5fd57423e142f2f8afc5a77e653fd6ed core/src/main/scala/kafka/tools/MirrorMaker.scala e4d1a86f673f7df7fe71ce44e3550fcde8d01bba perf/src/main/scala/kafka/perf/ProducerPerformance.scala 1490bdb3d52d5c901d7a46484935bda2f7ca system_test/utils/kafka_system_test_utils.py 423b512983c77b41b819912fe67b375a78faf644 Diff: https://reviews.apache.org/r/20997/diff/ Testing --- Thanks, Guozhang Wang
[jira] Subscription: outstanding kafka patches
Issue Subscription Filter: outstanding kafka patches (86 issues) The list of outstanding kafka patches Subscriber: kafka-mailing-list Key Summary KAFKA-1432 Make num.producerThreads configurable on new MirrrorMaker https://issues.apache.org/jira/browse/KAFKA-1432 KAFKA-1397 delete topic is not working https://issues.apache.org/jira/browse/KAFKA-1397 KAFKA-1394 Ensure last segment isn't deleted on expiration when there are unflushed messages https://issues.apache.org/jira/browse/KAFKA-1394 KAFKA-1384 Log Broker state https://issues.apache.org/jira/browse/KAFKA-1384 KAFKA-1380 0.8.1.1 release candidate https://issues.apache.org/jira/browse/KAFKA-1380 KAFKA-1372 Upgrade to Gradle 1.10 https://issues.apache.org/jira/browse/KAFKA-1372 KAFKA-1367 Broker topic metadata not kept in sync with ZooKeeper https://issues.apache.org/jira/browse/KAFKA-1367 KAFKA-1351 String.format is very expensive in Scala https://issues.apache.org/jira/browse/KAFKA-1351 KAFKA-1343 Kafka consumer iterator thread stalls https://issues.apache.org/jira/browse/KAFKA-1343 KAFKA-1324 Debian packaging https://issues.apache.org/jira/browse/KAFKA-1324 KAFKA-1308 Publish jar of test utilities to Maven https://issues.apache.org/jira/browse/KAFKA-1308 KAFKA-1303 metadata request in the new producer can be delayed https://issues.apache.org/jira/browse/KAFKA-1303 KAFKA-1300 Added WaitForReplaction admin tool. https://issues.apache.org/jira/browse/KAFKA-1300 KAFKA-1235 Enable server to indefinitely retry on controlled shutdown https://issues.apache.org/jira/browse/KAFKA-1235 KAFKA-1234 All kafka-run-class.sh to source in user config file (to set env vars like KAFKA_OPTS) https://issues.apache.org/jira/browse/KAFKA-1234 KAFKA-1230 shell script files under bin don't work with cygwin (bash on windows) https://issues.apache.org/jira/browse/KAFKA-1230 KAFKA-1215 Rack-Aware replica assignment option https://issues.apache.org/jira/browse/KAFKA-1215 KAFKA-1207 Launch Kafka from within Apache Mesos https://issues.apache.org/jira/browse/KAFKA-1207 KAFKA-1206 allow Kafka to start from a resource negotiator system https://issues.apache.org/jira/browse/KAFKA-1206 KAFKA-1194 The kafka broker cannot delete the old log files after the configured time https://issues.apache.org/jira/browse/KAFKA-1194 KAFKA-1190 create a draw performance graph script https://issues.apache.org/jira/browse/KAFKA-1190 KAFKA-1180 WhiteList topic filter gets a NullPointerException on complex Regex https://issues.apache.org/jira/browse/KAFKA-1180 KAFKA-1173 Using Vagrant to get up and running with Apache Kafka https://issues.apache.org/jira/browse/KAFKA-1173 KAFKA-1147 Consumer socket timeout should be greater than fetch max wait https://issues.apache.org/jira/browse/KAFKA-1147 KAFKA-1145 Broker fail to sync after restart https://issues.apache.org/jira/browse/KAFKA-1145 KAFKA-1144 commitOffsets can be passed the offsets to commit https://issues.apache.org/jira/browse/KAFKA-1144 KAFKA-1130 log.dirs is a confusing property name https://issues.apache.org/jira/browse/KAFKA-1130 KAFKA-1109 Need to fix GC log configuration code, not able to override KAFKA_GC_LOG_OPTS https://issues.apache.org/jira/browse/KAFKA-1109 KAFKA-1106 HighwaterMarkCheckpoint failure puting broker into a bad state https://issues.apache.org/jira/browse/KAFKA-1106 KAFKA-1093 Log.getOffsetsBefore(t, …) does not return the last confirmed offset before t https://issues.apache.org/jira/browse/KAFKA-1093 KAFKA-1086 Improve GetOffsetShell to find metadata automatically https://issues.apache.org/jira/browse/KAFKA-1086 KAFKA-1082 zkclient dies after UnknownHostException in zk reconnect https://issues.apache.org/jira/browse/KAFKA-1082 KAFKA-1049 Encoder implementations are required to provide an undocumented constructor. https://issues.apache.org/jira/browse/KAFKA-1049 KAFKA-1025 Producer.send should provide recoverability info on failiure https://issues.apache.org/jira/browse/KAFKA-1025 KAFKA-1011 Decompression and re-compression on MirrorMaker could result in messages being dropped in the pipeline https://issues.apache.org/jira/browse/KAFKA-1011 KAFKA-1005 kafka.perf.ConsumerPerformance not shutting down consumer https://issues.apache.org/jira/browse/KAFKA-1005 KAFKA-998 Producer should not retry on non-recoverable error codes https://issues.apache.org/jira/browse/KAFKA-998 KAFKA-997 Provide a strict verification mode when reading configuration properties https://issues.apache.org/jira/browse/KAFKA-997 KAFKA-996 Capitalize
Re: Review Request 20745: Patch for KAFKA-1397
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/20745/#review42195 --- core/src/test/scala/unit/kafka/admin/DeleteTopicTest.scala https://reviews.apache.org/r/20745/#comment75940 The follower could be the controller. So, we probably should shutdown a replica that's not the controller. - Jun Rao On May 5, 2014, 6:17 p.m., Timothy Chen wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/20745/ --- (Updated May 5, 2014, 6:17 p.m.) Review request for kafka. Bugs: KAFKA-1397 https://issues.apache.org/jira/browse/KAFKA-1397 Repository: kafka Description --- KAFKA-1397: Fix delete topic tests and deadlock Diffs - core/src/main/scala/kafka/admin/AdminUtils.scala 36ddeb44490e8343a4e8056c45726ac660e4b2f9 core/src/main/scala/kafka/controller/ControllerChannelManager.scala 919aeb26f93d2fc34d873cfb3dbfab7235a9d635 core/src/main/scala/kafka/controller/KafkaController.scala 933de9dd324c7086efe6aa610335ef370d9e9c12 core/src/main/scala/kafka/controller/ReplicaStateMachine.scala 0e47dac8cbf65a86d053a3371a18af467afd70ae core/src/main/scala/kafka/controller/TopicDeletionManager.scala e4bc2439ce1933c7c7571d255464ee678226a6cb core/src/main/scala/kafka/log/LogManager.scala ac67f081e6219fd2181479e7a2bb88ea6044e6cc core/src/main/scala/kafka/server/ReplicaManager.scala 11c20cee83fda9a492156674d351a0096b13fd99 core/src/main/scala/kafka/utils/ShutdownableThread.scala cf8adc9f468f4d6f01d1303efe39a3ec6f3d9b53 core/src/test/scala/unit/kafka/admin/DeleteTopicTest.scala 9c29e144bba2c9bafa91941b6ca5c263490693b3 core/src/test/scala/unit/kafka/server/DynamicConfigChangeTest.scala 5c487968014b56490eb2bc876cef1c52efd1cdad core/src/test/scala/unit/kafka/server/ServerShutdownTest.scala 014e9644dba1d65142f6a9abc745858870a46230 core/src/test/scala/unit/kafka/utils/TestUtils.scala 00bfba41fe008fe72c17fb788e3cada3daf8eb30 Diff: https://reviews.apache.org/r/20745/diff/ Testing --- Thanks, Timothy Chen
Re: Review Request 20718: Patch for KAFKA-1384
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/20718/#review42197 --- Compilation error when trying to run the unit tests. kafka_gradle/core/src/test/scala/unit/kafka/server/HighwatermarkPersistenceTest.scala:35: not enough arguments for constructor LogManager: (logDirs: Array[java.io.File],topicConfigs: scala.collection.Map[String,kafka.log.LogConfig],defaultConfig: kafka.log.LogConfig,cleanerConfig: kafka.log.CleanerConfig,flushCheckMs: Long,flushCheckpointMs: Long,retentionCheckMs: Long,scheduler: kafka.utils.Scheduler,brokerState: kafka.server.BrokerState,time: kafka.utils.Time)kafka.log.LogManager. Unspecified value parameter brokerState. val logManagers = configs.map(config = new LogManager(logDirs = config.logDirs.map(new File(_)).toArray, ^ /Users/jrao/Intellij/kafka_gradle/core/src/test/scala/unit/kafka/server/HighwatermarkPersistenceTest.scala:47: value logDirs is not a member of Nothing for(manager - logManagers; dir - manager.logDirs) ^ /Users/jrao/Intellij/kafka_gradle/core/src/test/scala/unit/kafka/server/HighwatermarkPersistenceTest.scala:67: value createLog is not a member of Nothing val log0 = logManagers(0).createLog(TopicAndPartition(topic, 0), LogConfig()) ^ /Users/jrao/Intellij/kafka_gradle/core/src/test/scala/unit/kafka/server/HighwatermarkPersistenceTest.scala:106: value createLog is not a member of Nothing val topic1Log0 = logManagers(0).createLog(TopicAndPartition(topic1, 0), LogConfig()) ^ /Users/jrao/Intellij/kafka_gradle/core/src/test/scala/unit/kafka/server/HighwatermarkPersistenceTest.scala:122: value createLog is not a member of Nothing val topic2Log0 = logManagers(0).createLog(TopicAndPartition(topic2, 0), LogConfig()) ^ /Users/jrao/Intellij/kafka_gradle/core/src/test/scala/unit/kafka/server/ReplicaManagerTest.scala:64: not enough arguments for constructor LogManager: (logDirs: Array[java.io.File],topicConfigs: scala.collection.Map[String,kafka.log.LogConfig],defaultConfig: kafka.log.LogConfig,cleanerConfig: kafka.log.CleanerConfig,flushCheckMs: Long,flushCheckpointMs: Long,retentionCheckMs: Long,scheduler: kafka.utils.Scheduler,brokerState: kafka.server.BrokerState,time: kafka.utils.Time)kafka.log.LogManager. Unspecified value parameter brokerState. return new LogManager(logDirs, - Jun Rao On May 5, 2014, 6:05 p.m., Timothy Chen wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/20718/ --- (Updated May 5, 2014, 6:05 p.m.) Review request for kafka. Bugs: KAFKA-1384 https://issues.apache.org/jira/browse/KAFKA-1384 Repository: kafka Description --- KAFKA-1384: Logging kafka state metric Diffs - core/src/main/scala/kafka/controller/KafkaController.scala 933de9dd324c7086efe6aa610335ef370d9e9c12 core/src/main/scala/kafka/log/Log.scala b7bc5ffdecba7221b3d2e4bca2b0c703bc74fa12 core/src/main/scala/kafka/log/LogManager.scala ac67f081e6219fd2181479e7a2bb88ea6044e6cc core/src/main/scala/kafka/server/BrokerStates.scala PRE-CREATION core/src/main/scala/kafka/server/KafkaServer.scala c208f83bed7fb91f07fae42f2b66892e6d46fecc core/src/main/scala/kafka/server/KafkaServerStartable.scala acda52b801714bcc182edc0ced925f0e4b493fc1 Diff: https://reviews.apache.org/r/20718/diff/ Testing --- Thanks, Timothy Chen
Re: Review Request 20745: Patch for KAFKA-1397
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/20745/#review42199 --- core/src/test/scala/unit/kafka/admin/DeleteTopicTest.scala https://reviews.apache.org/r/20745/#comment75941 Makes sense! - Timothy Chen On May 5, 2014, 9 p.m., Timothy Chen wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/20745/ --- (Updated May 5, 2014, 9 p.m.) Review request for kafka. Bugs: KAFKA-1397 https://issues.apache.org/jira/browse/KAFKA-1397 Repository: kafka Description --- KAFKA-1397: Fix delete topic tests and deadlock Diffs - core/src/main/scala/kafka/admin/AdminUtils.scala 36ddeb44490e8343a4e8056c45726ac660e4b2f9 core/src/main/scala/kafka/controller/ControllerChannelManager.scala 919aeb26f93d2fc34d873cfb3dbfab7235a9d635 core/src/main/scala/kafka/controller/KafkaController.scala 933de9dd324c7086efe6aa610335ef370d9e9c12 core/src/main/scala/kafka/controller/ReplicaStateMachine.scala 0e47dac8cbf65a86d053a3371a18af467afd70ae core/src/main/scala/kafka/controller/TopicDeletionManager.scala e4bc2439ce1933c7c7571d255464ee678226a6cb core/src/main/scala/kafka/log/LogManager.scala ac67f081e6219fd2181479e7a2bb88ea6044e6cc core/src/main/scala/kafka/server/ReplicaManager.scala 11c20cee83fda9a492156674d351a0096b13fd99 core/src/main/scala/kafka/utils/ShutdownableThread.scala cf8adc9f468f4d6f01d1303efe39a3ec6f3d9b53 core/src/test/scala/unit/kafka/admin/DeleteTopicTest.scala 9c29e144bba2c9bafa91941b6ca5c263490693b3 core/src/test/scala/unit/kafka/server/DynamicConfigChangeTest.scala 5c487968014b56490eb2bc876cef1c52efd1cdad core/src/test/scala/unit/kafka/server/ServerShutdownTest.scala 014e9644dba1d65142f6a9abc745858870a46230 core/src/test/scala/unit/kafka/utils/TestUtils.scala 00bfba41fe008fe72c17fb788e3cada3daf8eb30 Diff: https://reviews.apache.org/r/20745/diff/ Testing --- Thanks, Timothy Chen
Re: Review Request 20745: Patch for KAFKA-1397
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/20745/ --- (Updated May 5, 2014, 9 p.m.) Review request for kafka. Bugs: KAFKA-1397 https://issues.apache.org/jira/browse/KAFKA-1397 Repository: kafka Description --- KAFKA-1397: Fix delete topic tests and deadlock Diffs (updated) - core/src/main/scala/kafka/admin/AdminUtils.scala 36ddeb44490e8343a4e8056c45726ac660e4b2f9 core/src/main/scala/kafka/controller/ControllerChannelManager.scala 919aeb26f93d2fc34d873cfb3dbfab7235a9d635 core/src/main/scala/kafka/controller/KafkaController.scala 933de9dd324c7086efe6aa610335ef370d9e9c12 core/src/main/scala/kafka/controller/ReplicaStateMachine.scala 0e47dac8cbf65a86d053a3371a18af467afd70ae core/src/main/scala/kafka/controller/TopicDeletionManager.scala e4bc2439ce1933c7c7571d255464ee678226a6cb core/src/main/scala/kafka/log/LogManager.scala ac67f081e6219fd2181479e7a2bb88ea6044e6cc core/src/main/scala/kafka/server/ReplicaManager.scala 11c20cee83fda9a492156674d351a0096b13fd99 core/src/main/scala/kafka/utils/ShutdownableThread.scala cf8adc9f468f4d6f01d1303efe39a3ec6f3d9b53 core/src/test/scala/unit/kafka/admin/DeleteTopicTest.scala 9c29e144bba2c9bafa91941b6ca5c263490693b3 core/src/test/scala/unit/kafka/server/DynamicConfigChangeTest.scala 5c487968014b56490eb2bc876cef1c52efd1cdad core/src/test/scala/unit/kafka/server/ServerShutdownTest.scala 014e9644dba1d65142f6a9abc745858870a46230 core/src/test/scala/unit/kafka/utils/TestUtils.scala 00bfba41fe008fe72c17fb788e3cada3daf8eb30 Diff: https://reviews.apache.org/r/20745/diff/ Testing --- Thanks, Timothy Chen
[jira] [Commented] (KAFKA-1397) delete topic is not working
[ https://issues.apache.org/jira/browse/KAFKA-1397?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=13989968#comment-13989968 ] Timothy Chen commented on KAFKA-1397: - Updated reviewboard https://reviews.apache.org/r/20745/ against branch origin/trunk delete topic is not working Key: KAFKA-1397 URL: https://issues.apache.org/jira/browse/KAFKA-1397 Project: Kafka Issue Type: Bug Affects Versions: 0.8.2 Reporter: Jun Rao Attachments: KAFKA-1397.patch, KAFKA-1397_2014-04-28_14:48:32.patch, KAFKA-1397_2014-04-28_17:08:49.patch, KAFKA-1397_2014-04-30_14:55:28.patch, KAFKA-1397_2014-05-01_15:53:57.patch, KAFKA-1397_2014-05-01_18:12:24.patch, KAFKA-1397_2014-05-02_13:38:02.patch, KAFKA-1397_2014-05-05_11:17:59.patch, KAFKA-1397_2014-05-05_14:00:29.patch All unit tests are disabled since they hang transiently (see details in KAFKA-1391). -- This message was sent by Atlassian JIRA (v6.2#6252)
Re: Review Request 20718: Patch for KAFKA-1384
On May 5, 2014, 8:51 p.m., Jun Rao wrote: Compilation error when trying to run the unit tests. kafka_gradle/core/src/test/scala/unit/kafka/server/HighwatermarkPersistenceTest.scala:35: not enough arguments for constructor LogManager: (logDirs: Array[java.io.File],topicConfigs: scala.collection.Map[String,kafka.log.LogConfig],defaultConfig: kafka.log.LogConfig,cleanerConfig: kafka.log.CleanerConfig,flushCheckMs: Long,flushCheckpointMs: Long,retentionCheckMs: Long,scheduler: kafka.utils.Scheduler,brokerState: kafka.server.BrokerState,time: kafka.utils.Time)kafka.log.LogManager. Unspecified value parameter brokerState. val logManagers = configs.map(config = new LogManager(logDirs = config.logDirs.map(new File(_)).toArray, ^ /Users/jrao/Intellij/kafka_gradle/core/src/test/scala/unit/kafka/server/HighwatermarkPersistenceTest.scala:47: value logDirs is not a member of Nothing for(manager - logManagers; dir - manager.logDirs) ^ /Users/jrao/Intellij/kafka_gradle/core/src/test/scala/unit/kafka/server/HighwatermarkPersistenceTest.scala:67: value createLog is not a member of Nothing val log0 = logManagers(0).createLog(TopicAndPartition(topic, 0), LogConfig()) ^ /Users/jrao/Intellij/kafka_gradle/core/src/test/scala/unit/kafka/server/HighwatermarkPersistenceTest.scala:106: value createLog is not a member of Nothing val topic1Log0 = logManagers(0).createLog(TopicAndPartition(topic1, 0), LogConfig()) ^ /Users/jrao/Intellij/kafka_gradle/core/src/test/scala/unit/kafka/server/HighwatermarkPersistenceTest.scala:122: value createLog is not a member of Nothing val topic2Log0 = logManagers(0).createLog(TopicAndPartition(topic2, 0), LogConfig()) ^ /Users/jrao/Intellij/kafka_gradle/core/src/test/scala/unit/kafka/server/ReplicaManagerTest.scala:64: not enough arguments for constructor LogManager: (logDirs: Array[java.io.File],topicConfigs: scala.collection.Map[String,kafka.log.LogConfig],defaultConfig: kafka.log.LogConfig,cleanerConfig: kafka.log.CleanerConfig,flushCheckMs: Long,flushCheckpointMs: Long,retentionCheckMs: Long,scheduler: kafka.utils.Scheduler,brokerState: kafka.server.BrokerState,time: kafka.utils.Time)kafka.log.LogManager. Unspecified value parameter brokerState. return new LogManager(logDirs, Ah sorry, oddly I was able to run it but didn't try out all the tests. Fixed it in the next patch. - Timothy --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/20718/#review42197 --- On May 5, 2014, 9:25 p.m., Timothy Chen wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/20718/ --- (Updated May 5, 2014, 9:25 p.m.) Review request for kafka. Bugs: KAFKA-1384 https://issues.apache.org/jira/browse/KAFKA-1384 Repository: kafka Description --- KAFKA-1384: Logging kafka state metric Diffs - core/src/main/scala/kafka/controller/KafkaController.scala 933de9dd324c7086efe6aa610335ef370d9e9c12 core/src/main/scala/kafka/log/Log.scala b7bc5ffdecba7221b3d2e4bca2b0c703bc74fa12 core/src/main/scala/kafka/log/LogManager.scala ac67f081e6219fd2181479e7a2bb88ea6044e6cc core/src/main/scala/kafka/server/BrokerStates.scala PRE-CREATION core/src/main/scala/kafka/server/KafkaServer.scala c208f83bed7fb91f07fae42f2b66892e6d46fecc core/src/main/scala/kafka/server/KafkaServerStartable.scala acda52b801714bcc182edc0ced925f0e4b493fc1 core/src/test/scala/unit/kafka/log/LogManagerTest.scala be1a1ee3ac5c1dd8d658deabcfc740e88e053288 core/src/test/scala/unit/kafka/server/HighwatermarkPersistenceTest.scala a78f7cfa025b0c5d968381b6d5dc7aec894a612b core/src/test/scala/unit/kafka/server/ReplicaManagerTest.scala 41ebc7acae2efdd8a58964aa746c90fa71ac9ba4 Diff: https://reviews.apache.org/r/20718/diff/ Testing --- Thanks, Timothy Chen
[jira] [Commented] (KAFKA-1384) Log Broker state
[ https://issues.apache.org/jira/browse/KAFKA-1384?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=13989994#comment-13989994 ] Timothy Chen commented on KAFKA-1384: - Updated reviewboard https://reviews.apache.org/r/20718/ against branch origin/trunk Log Broker state - Key: KAFKA-1384 URL: https://issues.apache.org/jira/browse/KAFKA-1384 Project: Kafka Issue Type: New Feature Reporter: Timothy Chen Assignee: Timothy Chen Attachments: KAFKA-1384.patch, KAFKA-1384_2014-04-26_00:09:56.patch, KAFKA-1384_2014-05-01_18:47:03.patch, KAFKA-1384_2014-05-02_12:13:24.patch, KAFKA-1384_2014-05-05_11:05:15.patch, KAFKA-1384_2014-05-05_14:25:25.patch Currently we don't have visibility into what state the broker is currently in, ie: Startup - Running - Waiting Controlled shutdown - Shutting down So without knowing what state the broker it is it's hard to figure out what the current broker is performing. This ticket is to add a new metric to expose the current broker state. -- This message was sent by Atlassian JIRA (v6.2#6252)
Re: Review Request 20997: Fix KAFKA-1432: address Jun's comments
On May 2, 2014, 5:51 p.m., Neha Narkhede wrote: core/src/main/scala/kafka/producer/TransitProducer.scala, lines 24-25 https://reviews.apache.org/r/20997/diff/1/?file=573268#file573268line24 +1 on renaming this as per Jun's suggestion. If this is only for the tools, then let's move it to the tools package. Guozhang Wang wrote: ConsoleProducer is in kafka.producer ProducerPerformance is in kafka.perf MirrorMaker is in kafka.tools So I decided to put the BaseProducer in kafka.producer. Let me know if you have another preference. Neha Narkhede wrote: ConsoleProducer is a tool and should be moved to kafka.tools ProducerPerformance is also supposed to be in kafka.tools. It seems to me that it's better to do these moves and also move BaseProducer to kafka.tools. Agreed, but I would suggest we do this migration in another JIRA, where we can: 1. Move Consumer/ProducerPerformance and SimpleConsumerPerformance to tools and remove the perf sub-project. 2. Move ConsoleConsumer from kafka.consumer to kafka.tools. 3. Move other consumer related tools from kafka.consumer to kafka.tools. - Guozhang --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/20997/#review42028 --- On May 5, 2014, 5:31 p.m., Guozhang Wang wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/20997/ --- (Updated May 5, 2014, 5:31 p.m.) Review request for kafka. Bugs: KAFKA-1432 https://issues.apache.org/jira/browse/KAFKA-1432 Repository: kafka Description --- Address Jun's comments Add new TransitProducer KAFKA-1432.v1 Diffs - core/src/main/scala/kafka/producer/BaseProducer.scala PRE-CREATION core/src/main/scala/kafka/producer/ByteArrayPartitioner.scala 988e4374d8c9b30c969741c30c75f0c44f2c core/src/main/scala/kafka/producer/ConsoleProducer.scala b19ab49c5fd57423e142f2f8afc5a77e653fd6ed core/src/main/scala/kafka/tools/MirrorMaker.scala e4d1a86f673f7df7fe71ce44e3550fcde8d01bba perf/src/main/scala/kafka/perf/ProducerPerformance.scala 1490bdb3d52d5c901d7a46484935bda2f7ca system_test/utils/kafka_system_test_utils.py 423b512983c77b41b819912fe67b375a78faf644 Diff: https://reviews.apache.org/r/20997/diff/ Testing --- Thanks, Guozhang Wang
Re: Review Request 20997: Fix KAFKA-1432: address Neha's comments
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/20997/ --- (Updated May 5, 2014, 9:33 p.m.) Review request for kafka. Summary (updated) - Fix KAFKA-1432: address Neha's comments Bugs: KAFKA-1432 https://issues.apache.org/jira/browse/KAFKA-1432 Repository: kafka Description (updated) --- Address Neha's comments Address Jun's comments Add new TransitProducer KAFKA-1432.v1 Diffs (updated) - core/src/main/scala/kafka/producer/BaseProducer.scala PRE-CREATION core/src/main/scala/kafka/producer/ByteArrayPartitioner.scala 988e4374d8c9b30c969741c30c75f0c44f2c core/src/main/scala/kafka/producer/ConsoleProducer.scala b19ab49c5fd57423e142f2f8afc5a77e653fd6ed core/src/main/scala/kafka/tools/MirrorMaker.scala e4d1a86f673f7df7fe71ce44e3550fcde8d01bba core/src/main/scala/kafka/tools/newproducer/MirrorMaker.scala a969a22c6b82ed5c9f053a50594b1c7a768efd0e perf/src/main/scala/kafka/perf/ProducerPerformance.scala 1490bdb3d52d5c901d7a46484935bda2f7ca system_test/utils/kafka_system_test_utils.py 423b512983c77b41b819912fe67b375a78faf644 Diff: https://reviews.apache.org/r/20997/diff/ Testing --- Thanks, Guozhang Wang
[jira] [Updated] (KAFKA-1432) Make num.producerThreads configurable on new MirrrorMaker
[ https://issues.apache.org/jira/browse/KAFKA-1432?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Guozhang Wang updated KAFKA-1432: - Attachment: KAFKA-1432_2014-05-05_14:33:01.patch Make num.producerThreads configurable on new MirrrorMaker - Key: KAFKA-1432 URL: https://issues.apache.org/jira/browse/KAFKA-1432 Project: Kafka Issue Type: Bug Reporter: Guozhang Wang Assignee: Guozhang Wang Fix For: 0.8.2 Attachments: KAFKA-1432.patch, KAFKA-1432_2014-05-05_10:31:42.patch, KAFKA-1432_2014-05-05_14:33:01.patch Originally we make the num.producerThreads coupled with num.consumerThreads since we thought with the new producer, a single or couple of instances are sufficient for throughput. However, with compression turned on, which is now executed in the caller thread we still need configurable number of producer threads. -- This message was sent by Atlassian JIRA (v6.2#6252)
[jira] [Commented] (KAFKA-1432) Make num.producerThreads configurable on new MirrrorMaker
[ https://issues.apache.org/jira/browse/KAFKA-1432?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=13989997#comment-13989997 ] Guozhang Wang commented on KAFKA-1432: -- Updated reviewboard https://reviews.apache.org/r/20997/ against branch origin/trunk Make num.producerThreads configurable on new MirrrorMaker - Key: KAFKA-1432 URL: https://issues.apache.org/jira/browse/KAFKA-1432 Project: Kafka Issue Type: Bug Reporter: Guozhang Wang Assignee: Guozhang Wang Fix For: 0.8.2 Attachments: KAFKA-1432.patch, KAFKA-1432_2014-05-05_10:31:42.patch, KAFKA-1432_2014-05-05_14:33:01.patch Originally we make the num.producerThreads coupled with num.consumerThreads since we thought with the new producer, a single or couple of instances are sufficient for throughput. However, with compression turned on, which is now executed in the caller thread we still need configurable number of producer threads. -- This message was sent by Atlassian JIRA (v6.2#6252)
Re: Review Request 20997: Fix KAFKA-1432: address Neha's comments
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/20997/#review42212 --- core/src/main/scala/kafka/tools/MirrorMaker.scala https://reviews.apache.org/r/20997/#comment75975 This needs to change as well right? Maybe something like mirrormaker-consumer core/src/main/scala/kafka/tools/MirrorMaker.scala https://reviews.apache.org/r/20997/#comment75976 For consistency, this should be mirrormaker-producer - Neha Narkhede On May 5, 2014, 9:33 p.m., Guozhang Wang wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/20997/ --- (Updated May 5, 2014, 9:33 p.m.) Review request for kafka. Bugs: KAFKA-1432 https://issues.apache.org/jira/browse/KAFKA-1432 Repository: kafka Description --- Address Neha's comments Address Jun's comments Add new TransitProducer KAFKA-1432.v1 Diffs - core/src/main/scala/kafka/producer/BaseProducer.scala PRE-CREATION core/src/main/scala/kafka/producer/ByteArrayPartitioner.scala 988e4374d8c9b30c969741c30c75f0c44f2c core/src/main/scala/kafka/producer/ConsoleProducer.scala b19ab49c5fd57423e142f2f8afc5a77e653fd6ed core/src/main/scala/kafka/tools/MirrorMaker.scala e4d1a86f673f7df7fe71ce44e3550fcde8d01bba core/src/main/scala/kafka/tools/newproducer/MirrorMaker.scala a969a22c6b82ed5c9f053a50594b1c7a768efd0e perf/src/main/scala/kafka/perf/ProducerPerformance.scala 1490bdb3d52d5c901d7a46484935bda2f7ca system_test/utils/kafka_system_test_utils.py 423b512983c77b41b819912fe67b375a78faf644 Diff: https://reviews.apache.org/r/20997/diff/ Testing --- Thanks, Guozhang Wang
[jira] [Commented] (KAFKA-1432) Make num.producerThreads configurable on new MirrrorMaker
[ https://issues.apache.org/jira/browse/KAFKA-1432?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=13990052#comment-13990052 ] Guozhang Wang commented on KAFKA-1432: -- Updated reviewboard https://reviews.apache.org/r/20997/ against branch origin/trunk Make num.producerThreads configurable on new MirrrorMaker - Key: KAFKA-1432 URL: https://issues.apache.org/jira/browse/KAFKA-1432 Project: Kafka Issue Type: Bug Reporter: Guozhang Wang Assignee: Guozhang Wang Fix For: 0.8.2 Attachments: KAFKA-1432.patch, KAFKA-1432_2014-05-05_10:31:42.patch, KAFKA-1432_2014-05-05_14:33:01.patch, KAFKA-1432_2014-05-05_15:24:08.patch Originally we make the num.producerThreads coupled with num.consumerThreads since we thought with the new producer, a single or couple of instances are sufficient for throughput. However, with compression turned on, which is now executed in the caller thread we still need configurable number of producer threads. -- This message was sent by Atlassian JIRA (v6.2#6252)
Re: Review Request 20997: Fix KAFKA-1432: address Neha's comments
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/20997/ --- (Updated May 5, 2014, 10:24 p.m.) Review request for kafka. Bugs: KAFKA-1432 https://issues.apache.org/jira/browse/KAFKA-1432 Repository: kafka Description (updated) --- Address Neha's comments round two Address Neha's comments Address Jun's comments Add new TransitProducer KAFKA-1432.v1 Diffs (updated) - core/src/main/scala/kafka/producer/BaseProducer.scala PRE-CREATION core/src/main/scala/kafka/producer/ByteArrayPartitioner.scala 988e4374d8c9b30c969741c30c75f0c44f2c core/src/main/scala/kafka/producer/ConsoleProducer.scala b19ab49c5fd57423e142f2f8afc5a77e653fd6ed core/src/main/scala/kafka/tools/MirrorMaker.scala e4d1a86f673f7df7fe71ce44e3550fcde8d01bba core/src/main/scala/kafka/tools/newproducer/MirrorMaker.scala a969a22c6b82ed5c9f053a50594b1c7a768efd0e perf/src/main/scala/kafka/perf/ProducerPerformance.scala 1490bdb3d52d5c901d7a46484935bda2f7ca system_test/utils/kafka_system_test_utils.py 423b512983c77b41b819912fe67b375a78faf644 Diff: https://reviews.apache.org/r/20997/diff/ Testing --- Thanks, Guozhang Wang
Re: Review Request 20718: Patch for KAFKA-1384
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/20718/#review42218 --- core/src/test/scala/unit/kafka/log/LogManagerTest.scala https://reviews.apache.org/r/20718/#comment76008 Could we explicitly specify all parameters as we do in testRecoveryDirectoryMappingWithTrailingSlash()? core/src/test/scala/unit/kafka/log/LogManagerTest.scala https://reviews.apache.org/r/20718/#comment76009 Could we explicitly specify all parameters as we do in testRecoveryDirectoryMappingWithTrailingSlash()? core/src/test/scala/unit/kafka/log/LogManagerTest.scala https://reviews.apache.org/r/20718/#comment76010 Could we explicitly specify all parameters as we do in testRecoveryDirectoryMappingWithTrailingSlash()? - Jun Rao On May 5, 2014, 9:25 p.m., Timothy Chen wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/20718/ --- (Updated May 5, 2014, 9:25 p.m.) Review request for kafka. Bugs: KAFKA-1384 https://issues.apache.org/jira/browse/KAFKA-1384 Repository: kafka Description --- KAFKA-1384: Logging kafka state metric Diffs - core/src/main/scala/kafka/controller/KafkaController.scala 933de9dd324c7086efe6aa610335ef370d9e9c12 core/src/main/scala/kafka/log/Log.scala b7bc5ffdecba7221b3d2e4bca2b0c703bc74fa12 core/src/main/scala/kafka/log/LogManager.scala ac67f081e6219fd2181479e7a2bb88ea6044e6cc core/src/main/scala/kafka/server/BrokerStates.scala PRE-CREATION core/src/main/scala/kafka/server/KafkaServer.scala c208f83bed7fb91f07fae42f2b66892e6d46fecc core/src/main/scala/kafka/server/KafkaServerStartable.scala acda52b801714bcc182edc0ced925f0e4b493fc1 core/src/test/scala/unit/kafka/log/LogManagerTest.scala be1a1ee3ac5c1dd8d658deabcfc740e88e053288 core/src/test/scala/unit/kafka/server/HighwatermarkPersistenceTest.scala a78f7cfa025b0c5d968381b6d5dc7aec894a612b core/src/test/scala/unit/kafka/server/ReplicaManagerTest.scala 41ebc7acae2efdd8a58964aa746c90fa71ac9ba4 Diff: https://reviews.apache.org/r/20718/diff/ Testing --- Thanks, Timothy Chen
Re: Review Request 20997: Fix KAFKA-1432: address Neha's comments
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/20997/#review42221 --- Ship it! Ship It! - Neha Narkhede On May 5, 2014, 10:24 p.m., Guozhang Wang wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/20997/ --- (Updated May 5, 2014, 10:24 p.m.) Review request for kafka. Bugs: KAFKA-1432 https://issues.apache.org/jira/browse/KAFKA-1432 Repository: kafka Description --- Address Neha's comments round two Address Neha's comments Address Jun's comments Add new TransitProducer KAFKA-1432.v1 Diffs - core/src/main/scala/kafka/producer/BaseProducer.scala PRE-CREATION core/src/main/scala/kafka/producer/ByteArrayPartitioner.scala 988e4374d8c9b30c969741c30c75f0c44f2c core/src/main/scala/kafka/producer/ConsoleProducer.scala b19ab49c5fd57423e142f2f8afc5a77e653fd6ed core/src/main/scala/kafka/tools/MirrorMaker.scala e4d1a86f673f7df7fe71ce44e3550fcde8d01bba core/src/main/scala/kafka/tools/newproducer/MirrorMaker.scala a969a22c6b82ed5c9f053a50594b1c7a768efd0e perf/src/main/scala/kafka/perf/ProducerPerformance.scala 1490bdb3d52d5c901d7a46484935bda2f7ca system_test/utils/kafka_system_test_utils.py 423b512983c77b41b819912fe67b375a78faf644 Diff: https://reviews.apache.org/r/20997/diff/ Testing --- Thanks, Guozhang Wang
[jira] [Resolved] (KAFKA-1432) Make num.producerThreads configurable on new MirrrorMaker
[ https://issues.apache.org/jira/browse/KAFKA-1432?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Neha Narkhede resolved KAFKA-1432. -- Resolution: Fixed Thanks for the patches, Guozhang. Pushed to trunk Make num.producerThreads configurable on new MirrrorMaker - Key: KAFKA-1432 URL: https://issues.apache.org/jira/browse/KAFKA-1432 Project: Kafka Issue Type: Bug Reporter: Guozhang Wang Assignee: Guozhang Wang Fix For: 0.8.2 Attachments: KAFKA-1432.patch, KAFKA-1432_2014-05-05_10:31:42.patch, KAFKA-1432_2014-05-05_14:33:01.patch, KAFKA-1432_2014-05-05_15:24:08.patch Originally we make the num.producerThreads coupled with num.consumerThreads since we thought with the new producer, a single or couple of instances are sufficient for throughput. However, with compression turned on, which is now executed in the caller thread we still need configurable number of producer threads. -- This message was sent by Atlassian JIRA (v6.2#6252)
[jira] [Updated] (KAFKA-1431) ConsoleConsumer - Option to clean zk consumer path
[ https://issues.apache.org/jira/browse/KAFKA-1431?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] sriharsha chintalapani updated KAFKA-1431: -- Attachment: KAFKA-1431-v1.patch ConsoleConsumer - Option to clean zk consumer path -- Key: KAFKA-1431 URL: https://issues.apache.org/jira/browse/KAFKA-1431 Project: Kafka Issue Type: Bug Components: consumer Affects Versions: 0.8.1 Environment: All Reporter: Jeremy A Laycock Priority: Minor Labels: newbie Attachments: KAFKA-1431-v1.patch Raised in response to KAFKA-1426. Currently option from-beginning auto deletes the zk consumer path. This is confusing and un-expected behaviour. Suggest a separate option to clean the console consumer path. -- This message was sent by Atlassian JIRA (v6.2#6252)
[jira] [Updated] (KAFKA-1431) ConsoleConsumer - Option to clean zk consumer path
[ https://issues.apache.org/jira/browse/KAFKA-1431?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] sriharsha chintalapani updated KAFKA-1431: -- Status: Patch Available (was: Open) ConsoleConsumer - Option to clean zk consumer path -- Key: KAFKA-1431 URL: https://issues.apache.org/jira/browse/KAFKA-1431 Project: Kafka Issue Type: Bug Components: consumer Affects Versions: 0.8.1 Environment: All Reporter: Jeremy A Laycock Priority: Minor Labels: newbie Attachments: KAFKA-1431-v1.patch Raised in response to KAFKA-1426. Currently option from-beginning auto deletes the zk consumer path. This is confusing and un-expected behaviour. Suggest a separate option to clean the console consumer path. -- This message was sent by Atlassian JIRA (v6.2#6252)
[jira] [Commented] (KAFKA-1384) Log Broker state
[ https://issues.apache.org/jira/browse/KAFKA-1384?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=13990123#comment-13990123 ] Timothy Chen commented on KAFKA-1384: - Updated reviewboard https://reviews.apache.org/r/20718/ against branch origin/trunk Log Broker state - Key: KAFKA-1384 URL: https://issues.apache.org/jira/browse/KAFKA-1384 Project: Kafka Issue Type: New Feature Reporter: Timothy Chen Assignee: Timothy Chen Attachments: KAFKA-1384.patch, KAFKA-1384_2014-04-26_00:09:56.patch, KAFKA-1384_2014-05-01_18:47:03.patch, KAFKA-1384_2014-05-02_12:13:24.patch, KAFKA-1384_2014-05-05_11:05:15.patch, KAFKA-1384_2014-05-05_14:25:25.patch, KAFKA-1384_2014-05-05_17:14:57.patch Currently we don't have visibility into what state the broker is currently in, ie: Startup - Running - Waiting Controlled shutdown - Shutting down So without knowing what state the broker it is it's hard to figure out what the current broker is performing. This ticket is to add a new metric to expose the current broker state. -- This message was sent by Atlassian JIRA (v6.2#6252)
[jira] [Updated] (KAFKA-1384) Log Broker state
[ https://issues.apache.org/jira/browse/KAFKA-1384?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Timothy Chen updated KAFKA-1384: Attachment: KAFKA-1384_2014-05-05_17:14:57.patch Log Broker state - Key: KAFKA-1384 URL: https://issues.apache.org/jira/browse/KAFKA-1384 Project: Kafka Issue Type: New Feature Reporter: Timothy Chen Assignee: Timothy Chen Attachments: KAFKA-1384.patch, KAFKA-1384_2014-04-26_00:09:56.patch, KAFKA-1384_2014-05-01_18:47:03.patch, KAFKA-1384_2014-05-02_12:13:24.patch, KAFKA-1384_2014-05-05_11:05:15.patch, KAFKA-1384_2014-05-05_14:25:25.patch, KAFKA-1384_2014-05-05_17:14:57.patch Currently we don't have visibility into what state the broker is currently in, ie: Startup - Running - Waiting Controlled shutdown - Shutting down So without knowing what state the broker it is it's hard to figure out what the current broker is performing. This ticket is to add a new metric to expose the current broker state. -- This message was sent by Atlassian JIRA (v6.2#6252)
Re: Review Request 20718: Patch for KAFKA-1384
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/20718/ --- (Updated May 6, 2014, 12:15 a.m.) Review request for kafka. Bugs: KAFKA-1384 https://issues.apache.org/jira/browse/KAFKA-1384 Repository: kafka Description --- KAFKA-1384: Logging kafka state metric Diffs (updated) - core/src/main/scala/kafka/controller/KafkaController.scala 933de9dd324c7086efe6aa610335ef370d9e9c12 core/src/main/scala/kafka/log/Log.scala b7bc5ffdecba7221b3d2e4bca2b0c703bc74fa12 core/src/main/scala/kafka/log/LogManager.scala ac67f081e6219fd2181479e7a2bb88ea6044e6cc core/src/main/scala/kafka/server/BrokerStates.scala PRE-CREATION core/src/main/scala/kafka/server/KafkaServer.scala c208f83bed7fb91f07fae42f2b66892e6d46fecc core/src/main/scala/kafka/server/KafkaServerStartable.scala acda52b801714bcc182edc0ced925f0e4b493fc1 core/src/test/scala/unit/kafka/log/LogManagerTest.scala be1a1ee3ac5c1dd8d658deabcfc740e88e053288 core/src/test/scala/unit/kafka/server/HighwatermarkPersistenceTest.scala a78f7cfa025b0c5d968381b6d5dc7aec894a612b core/src/test/scala/unit/kafka/server/ReplicaManagerTest.scala 41ebc7acae2efdd8a58964aa746c90fa71ac9ba4 Diff: https://reviews.apache.org/r/20718/diff/ Testing --- Thanks, Timothy Chen
Re: Review for the new consumer APIs
Hi Neha, How will new Consumer help us with implementing following use case? We have heartbeat as one of topics and all application servers publish metric to this topic. We have to meet near real-time consume SLA (less than 30 seconds). 1) We would like to find out what is latest message per partition that current consumer is connected? 2) If the consumer lags behind by certain offset or by time, consumer can seek to particular offset(which we can use seek method for this). 3) How can we start a temp consumer for same partition to read messages based on offset range (last consume offset from part 2 to current offset that we jumped to in part 2) ? Basically, is there a QOS concept per partition where consumer always needs to consume latest message and detect a lag behind and start TEMP consumer for back-fill. How does Linked in handle the near real time consumption for operation metrics ? Thanks, Bhavesh On Sat, Apr 12, 2014 at 6:58 PM, Neha Narkhede neha.narkh...@gmail.comwrote: Why cant we pass a callback in subscribe itself? Mainly because it will make the processing kind of awkward since you need to access the other consumer APIs while processing the messages. Your suggestion does point out a problem with the poll() API though. Here is the initial proposal of the poll() API- ListConsumerRecord poll(long timeout, TimeUnit unit); The application subscribes to topics or partitions and expects to process messages per topic or per partition respectively. By just returning a list of ConsumerRecord objects, we make it difficult for the application to process messages naturally grouped by topic or partition. After some thought, I changed it to - MapString, ConsumerRecordMetadata poll(long timeout, TimeUnit unit); ConsumerRecordMetadata allows you to get records for a particular partition or get records for all partitions. The second change I made is to the commit APIs. To remain consistent with the Producer, I changed commit() to return a Future and got rid of commitAsync(). This will easily support the sync and async commit use cases. MapTopicPartition,OffsetMetadata http://people.apache.org/~nehanarkhede/kafka-0.9-consumer-javadoc/doc/org/apache/kafka/clients/consumer/OffsetMetadata.html commit(MapTopicPartition,Long offsets); I'm looking for feedback on these changes. I've published the new javadoc to the same location http://people.apache.org/~nehanarkhede/kafka-0.9-consumer-javadoc/doc. Appreciate if someone can take a look. Thanks, Neha On Tue, Apr 8, 2014 at 9:50 PM, pushkar priyadarshi priyadarshi.push...@gmail.com wrote: Was trying to understand when we have subscribe then why poll is a separate API.Why cant we pass a callback in subscribe itself? On Mon, Apr 7, 2014 at 9:51 PM, Neha Narkhede neha.narkh...@gmail.com wrote: Hi, I'm looking for people to review the new consumers APIs. Patch is posted at https://issues.apache.org/jira/browse/KAFKA-1328 Thanks, Neha
Re: Review Request 20718: Patch for KAFKA-1384
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/20718/#review42228 --- Ship it! Minor comment, mentioned earlier: prefer just State instead of BrokerState. - Joel Koshy On May 6, 2014, 12:15 a.m., Timothy Chen wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/20718/ --- (Updated May 6, 2014, 12:15 a.m.) Review request for kafka. Bugs: KAFKA-1384 https://issues.apache.org/jira/browse/KAFKA-1384 Repository: kafka Description --- KAFKA-1384: Logging kafka state metric Diffs - core/src/main/scala/kafka/controller/KafkaController.scala 933de9dd324c7086efe6aa610335ef370d9e9c12 core/src/main/scala/kafka/log/Log.scala b7bc5ffdecba7221b3d2e4bca2b0c703bc74fa12 core/src/main/scala/kafka/log/LogManager.scala ac67f081e6219fd2181479e7a2bb88ea6044e6cc core/src/main/scala/kafka/server/BrokerStates.scala PRE-CREATION core/src/main/scala/kafka/server/KafkaServer.scala c208f83bed7fb91f07fae42f2b66892e6d46fecc core/src/main/scala/kafka/server/KafkaServerStartable.scala acda52b801714bcc182edc0ced925f0e4b493fc1 core/src/test/scala/unit/kafka/log/LogManagerTest.scala be1a1ee3ac5c1dd8d658deabcfc740e88e053288 core/src/test/scala/unit/kafka/server/HighwatermarkPersistenceTest.scala a78f7cfa025b0c5d968381b6d5dc7aec894a612b core/src/test/scala/unit/kafka/server/ReplicaManagerTest.scala 41ebc7acae2efdd8a58964aa746c90fa71ac9ba4 Diff: https://reviews.apache.org/r/20718/diff/ Testing --- Thanks, Timothy Chen
[jira] [Commented] (KAFKA-1273) Brokers should make sure replica.fetch.max.bytes = message.max.bytes
[ https://issues.apache.org/jira/browse/KAFKA-1273?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=13990162#comment-13990162 ] sriharsha chintalapani commented on KAFKA-1273: --- I am not able to reproduce this in the trunk as kafka-server-start.sh fails to start the broker throwing java.lang.IllegalArgumentException: requirement failed: replica.fetch.max.bytes has value 1024 which is not in the range (2048,2147483647). its checking if the replica.fetch.max.bytes = message.max.bytes. Brokers should make sure replica.fetch.max.bytes = message.max.bytes - Key: KAFKA-1273 URL: https://issues.apache.org/jira/browse/KAFKA-1273 Project: Kafka Issue Type: Bug Components: replication Affects Versions: 0.8.0 Reporter: Dong Zhong Assignee: Neha Narkhede Labels: newbie If message.max.bytes is larger than replica.fetch.max.bytes,followers can't fetch data from the leader and will incur endless retry. And this may cause high network traffic between followers and leaders. Brokers should make sure replica.fetch.max.bytes = message.max.bytes by adding a sanity check, or throw an exception. -- This message was sent by Atlassian JIRA (v6.2#6252)
QOS on Producer Side
We are using Kafka for operation metrics and we do not want to loose any data at all if there is issue with Network or all brokers needs to be rebooted or operation reason down time while all producers are still producing data on front end side. We use async to publish messages and we are using Kafka version 0.8.0. Has any one implemented buffering on local disk (on producer side) and transmit messages when network connection is restored? How do I get handle to list of messages async thread could not transfer after x reties ? I know new producer API has callback interface, but is per message not per producer instance ? Is this final new Producer API ? http://empathybox.com/kafka-javadoc/index.html?kafka/clients/producer/KafkaProducer.html is there a plan to add method like. onFailure(ListMessages messages, Exception exception ) ? Basically, I have to address the QOS on producer side, and be able to buffer on disk and retransmit all message to partitions that are reserved for messages that happened in past. How does Linked-in handle QOS on producer side ? Is there any plan to add this QOS feature on Producer Side with Strategies to store and retransmit the message ? If we do get the list of messages is call back, will it be compressed data ? I would appreciate Kafka Developers and others feedback on how to implement QOS. Thanks, Bhavesh
Re: QOS on Producer Side
Thanks for answers. Does the callback get call on failure only or for success as well ? Also, how do I do this on Kafka 0.8.0 ? Is there any plan for adding buffering on disk for next version ? Also, when application restart, kafka producer will have to transmit the messages (buffered messages on disk) ? I can not upgrade to Kafka 0.8.1.1 right now. how can I achieve this on Kafka version 0.8.0. Also, when retransmitting the messages I wanted to transfer dedicated Kafka back-up or reserved partition for topic (near-real time topic). How would you handle retransmission of messages ? Any idea about implementation will be very helpful. Thanks, Bhavesh On Mon, May 5, 2014 at 9:29 PM, Jun Rao jun...@gmail.com wrote: At LinkedIn, we only buffer messages in memory in the producer. We try as much as we can to make the Kafka cluster always available (with replication). The on failure feature you mentioned can be implemented by providing a callback. You can see the example in the javadoc. *send http://empathybox.com/kafka-javadoc/kafka/clients/producer/KafkaProducer.html#send(kafka.clients.producer.ProducerRecord , kafka.clients.producer.Callback)*(ProducerRecord http://empathybox.com/kafka-javadoc/kafka/clients/producer/ProducerRecord.html record, Callback http://empathybox.com/kafka-javadoc/kafka/clients/producer/Callback.html callback) Thanks, Jun On Mon, May 5, 2014 at 6:30 PM, Bhavesh Mistry mistry.p.bhav...@gmail.comwrote: We are using Kafka for operation metrics and we do not want to loose any data at all if there is issue with Network or all brokers needs to be rebooted or operation reason down time while all producers are still producing data on front end side. We use async to publish messages and we are using Kafka version 0.8.0. Has any one implemented buffering on local disk (on producer side) and transmit messages when network connection is restored? How do I get handle to list of messages async thread could not transfer after x reties ? I know new producer API has callback interface, but is per message not per producer instance ? Is this final new Producer API ? http://empathybox.com/kafka-javadoc/index.html?kafka/clients/producer/KafkaProducer.html is there a plan to add method like. onFailure(ListMessages messages, Exception exception ) ? Basically, I have to address the QOS on producer side, and be able to buffer on disk and retransmit all message to partitions that are reserved for messages that happened in past. How does Linked-in handle QOS on producer side ? Is there any plan to add this QOS feature on Producer Side with Strategies to store and retransmit the message ? If we do get the list of messages is call back, will it be compressed data ? I would appreciate Kafka Developers and others feedback on how to implement QOS. Thanks, Bhavesh