[jira] [Commented] (KAFKA-1995) JMS to Kafka: Inbuilt JMSAdaptor/JMSProxy/JMSBridge (Client can speak JMS but hit Kafka)
[ https://issues.apache.org/jira/browse/KAFKA-1995?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14349493#comment-14349493 ] Rekha Joshi commented on KAFKA-1995: Great.Thanks for your reply, [~ewencp] [~joestein] ! JMS to Kafka: Inbuilt JMSAdaptor/JMSProxy/JMSBridge (Client can speak JMS but hit Kafka) Key: KAFKA-1995 URL: https://issues.apache.org/jira/browse/KAFKA-1995 Project: Kafka Issue Type: Wish Components: core Affects Versions: 0.8.3 Reporter: Rekha Joshi Kafka is a great alternative to JMS, providing high performance, throughput as scalable, distributed pub sub/commit log service. However there always exist traditional systems running on JMS. Rather than rewriting, it would be great if we just had an inbuilt JMSAdaptor/JMSProxy/JMSBridge by which client can speak JMS but hit Kafka behind-the-scene. Something like Chukwa's o.a.h.chukwa.datacollection.adaptor.jms.JMSAdaptor, which receives msg off JMS queue and transforms to a Chukwa chunk? I have come across folks talking of this need in past as well.Is it considered and/or part of the roadmap? http://grokbase.com/t/kafka/users/131cst8xpv/stomp-binding-for-kafka http://grokbase.com/t/kafka/users/148dm4247q/consuming-messages-from-kafka-and-pushing-on-to-a-jms-queue http://grokbase.com/t/kafka/users/143hjepbn2/request-kafka-zookeeper-jms-details Looking for inputs on correct way to approach this so to retain all good features of Kafka while still not rewriting entire application.Possible? -- This message was sent by Atlassian JIRA (v6.3.4#6332)
Re: Review Request 31650: Drag Coordinator and FetchManager out of KafkaConsumer, fix a bunch of consumer test issues
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/31650/ --- (Updated March 5, 2015, 10:55 p.m.) Review request for kafka. Bugs: KAFKA-1910 https://issues.apache.org/jira/browse/KAFKA-1910 Repository: kafka Description --- See comments in KAFKA-1910 Diffs (updated) - clients/src/main/java/org/apache/kafka/clients/CommonClientConfigs.java 06fcfe62cc1fe76f58540221698ef076fe150e96 clients/src/main/java/org/apache/kafka/clients/KafkaClient.java 8a3e55aaff7d8c26e56a8407166a4176c1da2644 clients/src/main/java/org/apache/kafka/clients/NetworkClient.java a7fa4a9dfbcfbc4d9e9259630253cbcded158064 clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerConfig.java 5fb21001abd77cac839bd724afa04e377a3e82aa clients/src/main/java/org/apache/kafka/clients/consumer/KafkaConsumer.java 67ceb754a52c07143c69b053fe128b9e24060b99 clients/src/main/java/org/apache/kafka/clients/consumer/internals/Coordinator.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/internals/FetchManager.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/internals/Heartbeat.java ee0751e4949120d114202c2299d49612a89b9d97 clients/src/main/java/org/apache/kafka/clients/consumer/internals/SubscriptionState.java d41d3068c11d4b5c640467dc0ae1b7c20a8d128c clients/src/main/java/org/apache/kafka/clients/producer/KafkaProducer.java 7397e565fd865214529ffccadd4222d835ac8110 clients/src/main/java/org/apache/kafka/clients/producer/ProducerConfig.java 122375c473bf73caf05299b9f5174c6b226ca863 clients/src/main/java/org/apache/kafka/clients/producer/internals/Sender.java ed9c63a6679e3aaf83d19fde19268553a4c107c2 clients/src/main/java/org/apache/kafka/common/network/Selector.java 6baad9366a1975dbaba1786da91efeaa38533319 clients/src/main/java/org/apache/kafka/common/protocol/Errors.java ad2171f5417c93194f5f234bdc7fdd0b8d59a8a8 clients/src/main/java/org/apache/kafka/common/record/MemoryRecords.java 083e7a39249ab56a73a014b106876244d619f189 clients/src/main/java/org/apache/kafka/common/requests/FetchResponse.java e67c4c8332cb1dd3d9cde5de687df7760045dfe6 clients/src/main/java/org/apache/kafka/common/requests/HeartbeatResponse.java 0057496228feeeccbc0c009a42f5268fa2cb8611 clients/src/main/java/org/apache/kafka/common/requests/JoinGroupRequest.java 8c50e9be534c61ecf56106bf2b68cf678ea50d66 clients/src/main/java/org/apache/kafka/common/requests/JoinGroupResponse.java 52b1803d8b558c1eeb978ba8821496c7d3c20a6b clients/src/main/java/org/apache/kafka/common/requests/ListOffsetResponse.java cfac47a4a05dc8a535595542d93e55237b7d1e93 clients/src/main/java/org/apache/kafka/common/requests/MetadataResponse.java 90f31413d7d80a06c0af359009cc271aa0c67be3 clients/src/main/java/org/apache/kafka/common/requests/OffsetCommitResponse.java 4d3b9ececee4b4c0b50ba99da2ddbbb15f9cc08d clients/src/main/java/org/apache/kafka/common/requests/OffsetFetchResponse.java edbed5880dc44fc178737a5e298c106a00f38443 clients/src/main/java/org/apache/kafka/common/requests/ProduceResponse.java a00dcdf15d1c7bac7228be140647bd7d849deb9b clients/src/test/java/org/apache/kafka/clients/MockClient.java 8f1a7a625e4eeafa44bbf9e5cff987de86c949be clients/src/test/java/org/apache/kafka/clients/consumer/internals/CoordinatorTest.java PRE-CREATION clients/src/test/java/org/apache/kafka/clients/consumer/internals/FetchManagerTest.java PRE-CREATION clients/src/test/java/org/apache/kafka/clients/consumer/internals/HeartbeatTest.java PRE-CREATION clients/src/test/java/org/apache/kafka/clients/consumer/internals/SubscriptionStateTest.java 090087a319e2697d3a0653ca947d2cfa6d53f6c2 clients/src/test/java/org/apache/kafka/clients/producer/internals/RecordAccumulatorTest.java c1bc40648479d4c2ae4ac52f40dadc070a6bcf6f clients/src/test/java/org/apache/kafka/clients/producer/internals/SenderTest.java ea56c997cb90d5bac8e3706dfc1eaae5b1555ccb clients/src/test/java/org/apache/kafka/common/record/MemoryRecordsTest.java e343327faf115a901657ec6da8e0c5b8bbf0b76a core/src/main/scala/kafka/common/ErrorMapping.scala eedc2f5f21dd8755fba891998456351622e17047 core/src/main/scala/kafka/common/NoOffsetsCommittedException.scala PRE-CREATION core/src/main/scala/kafka/common/OffsetMetadataAndError.scala 4cabffeacea09a49913505db19a96a55d58c0909 core/src/main/scala/kafka/coordinator/ConsumerCoordinator.scala 21790a5059ee00d6610be6f0389445327b88db1d core/src/main/scala/kafka/coordinator/ConsumerRegistry.scala b65c04d0a5d53bf92299d5f67f112be3da3bf77d core/src/main/scala/kafka/coordinator/DelayedHeartbeat.scala b1248e95d8a648b461f604c154879cc95dc7b1cb core/src/main/scala/kafka/coordinator/GroupRegistry.scala 7d17e102235134b6312271c4061abd27d7177f7e
Re: Review Request 31650: Drag Coordinator and FetchManager out of KafkaConsumer, fix a bunch of consumer test issues
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/31650/#review75406 --- clients/src/test/java/org/apache/kafka/clients/consumer/internals/CoordinatorTest.java https://reviews.apache.org/r/31650/#comment122464 I think these scenarios should be split up into separate tests. clients/src/test/java/org/apache/kafka/clients/consumer/internals/CoordinatorTest.java https://reviews.apache.org/r/31650/#comment122463 ditto clients/src/test/java/org/apache/kafka/clients/consumer/internals/CoordinatorTest.java https://reviews.apache.org/r/31650/#comment122465 ditto clients/src/test/java/org/apache/kafka/clients/consumer/internals/CoordinatorTest.java https://reviews.apache.org/r/31650/#comment122469 ditto clients/src/test/java/org/apache/kafka/clients/consumer/internals/FetchManagerTest.java https://reviews.apache.org/r/31650/#comment122470 ditto clients/src/test/java/org/apache/kafka/clients/consumer/internals/HeartbeatTest.java https://reviews.apache.org/r/31650/#comment122461 This class is missing a test where shouldHeartbeat should return false. clients/src/test/java/org/apache/kafka/clients/consumer/internals/HeartbeatTest.java https://reviews.apache.org/r/31650/#comment122443 Rather than add nondeterminism to the unit test with System.currentTimeMillis() and make it slower with the sleep call, we can just pass in longs. - Onur Karaman On March 5, 2015, 10:57 p.m., Guozhang Wang wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/31650/ --- (Updated March 5, 2015, 10:57 p.m.) Review request for kafka. Bugs: KAFKA-1910 https://issues.apache.org/jira/browse/KAFKA-1910 Repository: kafka Description --- See comments in KAFKA-1910; Updated RB includes unit test for Coordinator / FetchManager / Heartbeat and a couple changes on MemoryRecords and test utils. Diffs - clients/src/main/java/org/apache/kafka/clients/CommonClientConfigs.java 06fcfe62cc1fe76f58540221698ef076fe150e96 clients/src/main/java/org/apache/kafka/clients/KafkaClient.java 8a3e55aaff7d8c26e56a8407166a4176c1da2644 clients/src/main/java/org/apache/kafka/clients/NetworkClient.java a7fa4a9dfbcfbc4d9e9259630253cbcded158064 clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerConfig.java 5fb21001abd77cac839bd724afa04e377a3e82aa clients/src/main/java/org/apache/kafka/clients/consumer/KafkaConsumer.java 67ceb754a52c07143c69b053fe128b9e24060b99 clients/src/main/java/org/apache/kafka/clients/consumer/internals/Coordinator.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/internals/FetchManager.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/internals/Heartbeat.java ee0751e4949120d114202c2299d49612a89b9d97 clients/src/main/java/org/apache/kafka/clients/consumer/internals/SubscriptionState.java d41d3068c11d4b5c640467dc0ae1b7c20a8d128c clients/src/main/java/org/apache/kafka/clients/producer/KafkaProducer.java 7397e565fd865214529ffccadd4222d835ac8110 clients/src/main/java/org/apache/kafka/clients/producer/ProducerConfig.java 122375c473bf73caf05299b9f5174c6b226ca863 clients/src/main/java/org/apache/kafka/clients/producer/internals/Sender.java ed9c63a6679e3aaf83d19fde19268553a4c107c2 clients/src/main/java/org/apache/kafka/common/network/Selector.java 6baad9366a1975dbaba1786da91efeaa38533319 clients/src/main/java/org/apache/kafka/common/protocol/Errors.java ad2171f5417c93194f5f234bdc7fdd0b8d59a8a8 clients/src/main/java/org/apache/kafka/common/record/MemoryRecords.java 083e7a39249ab56a73a014b106876244d619f189 clients/src/main/java/org/apache/kafka/common/requests/FetchResponse.java e67c4c8332cb1dd3d9cde5de687df7760045dfe6 clients/src/main/java/org/apache/kafka/common/requests/HeartbeatResponse.java 0057496228feeeccbc0c009a42f5268fa2cb8611 clients/src/main/java/org/apache/kafka/common/requests/JoinGroupRequest.java 8c50e9be534c61ecf56106bf2b68cf678ea50d66 clients/src/main/java/org/apache/kafka/common/requests/JoinGroupResponse.java 52b1803d8b558c1eeb978ba8821496c7d3c20a6b clients/src/main/java/org/apache/kafka/common/requests/ListOffsetResponse.java cfac47a4a05dc8a535595542d93e55237b7d1e93 clients/src/main/java/org/apache/kafka/common/requests/MetadataResponse.java 90f31413d7d80a06c0af359009cc271aa0c67be3 clients/src/main/java/org/apache/kafka/common/requests/OffsetCommitResponse.java 4d3b9ececee4b4c0b50ba99da2ddbbb15f9cc08d clients/src/main/java/org/apache/kafka/common/requests/OffsetFetchResponse.java
Re: Review Request 31706: Patch for KAFKA-1997
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/31706/#review75409 --- Sorry this is not a thorough review but a first pass. I can dig deeper into it later. core/src/main/scala/kafka/consumer/PartitionAssignor.scala https://reviews.apache.org/r/31706/#comment122447 Why does this need to be a pool? i.e., rebalance is done while holding a lock. core/src/main/scala/kafka/consumer/PartitionAssignor.scala https://reviews.apache.org/r/31706/#comment122445 valueFactory = ... (named parameters make the code clearer) core/src/main/scala/kafka/consumer/PartitionAssignor.scala https://reviews.apache.org/r/31706/#comment122446 or even better zk consumer connector should just use a scala option type right? core/src/main/scala/kafka/consumer/ZookeeperConsumerConnector.scala https://reviews.apache.org/r/31706/#comment122460 pre-existing, but given the references to assignment I think assignment is more consistent (than ownership) core/src/main/scala/kafka/consumer/ZookeeperConsumerConnector.scala https://reviews.apache.org/r/31706/#comment122456 Rather build out the val here, can you move this to a separate line above? core/src/main/scala/kafka/consumer/ZookeeperConsumerConnector.scala https://reviews.apache.org/r/31706/#comment122457 Rather build out the val here, can you move this to a separate line above? Also, we can call it globalPartitionAssignment core/src/main/scala/kafka/consumer/ZookeeperConsumerConnector.scala https://reviews.apache.org/r/31706/#comment122458 partitionOwnerships - topicPartitionAssignment core/src/main/scala/kafka/tools/MirrorMaker.scala https://reviews.apache.org/r/31706/#comment122471 the same core/src/main/scala/kafka/tools/MirrorMaker.scala https://reviews.apache.org/r/31706/#comment122472 each mirror maker thread periodically flushes the producer and then commits all offsets core/src/main/scala/kafka/tools/MirrorMaker.scala https://reviews.apache.org/r/31706/#comment122475 Why was this change made? core/src/main/scala/kafka/tools/MirrorMaker.scala https://reviews.apache.org/r/31706/#comment122478 typo core/src/main/scala/kafka/tools/MirrorMaker.scala https://reviews.apache.org/r/31706/#comment122479 how does the user override? core/src/main/scala/kafka/tools/MirrorMaker.scala https://reviews.apache.org/r/31706/#comment122481 Could we use an explicit object to synchronize (as opposed to this)? core/src/main/scala/kafka/tools/MirrorMaker.scala https://reviews.apache.org/r/31706/#comment122482 rename to maybeFlushAndCommitOffsets - Joel Koshy On March 4, 2015, 11:42 p.m., Jiangjie Qin wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/31706/ --- (Updated March 4, 2015, 11:42 p.m.) Review request for kafka. Bugs: KAFKA-1997 https://issues.apache.org/jira/browse/KAFKA-1997 Repository: kafka Description --- Addressed Guozhang's comments. Changed the exit behavior on send failure because close(0) is not ready yet. Will submit followup patch after KAFKA-1660 is checked in. Expanded imports from _ and * to full class path Diffs - clients/src/main/java/org/apache/kafka/clients/producer/internals/RecordAccumulator.java d5c79e2481d5e9a2524ac2ef6a6879f61cb7cb5f core/src/main/scala/kafka/consumer/PartitionAssignor.scala e6ff7683a0df4a7d221e949767e57c34703d5aad core/src/main/scala/kafka/consumer/ZookeeperConsumerConnector.scala 5487259751ebe19f137948249aa1fd2637d2deb4 core/src/main/scala/kafka/javaapi/consumer/ConsumerRebalanceListener.java 7f45a90ba6676290172b7da54c15ee5dc1a42a2e core/src/main/scala/kafka/tools/MirrorMaker.scala 5374280dc97dc8e01e9b3ba61fd036dc13ae48cb core/src/test/scala/unit/kafka/consumer/PartitionAssignorTest.scala 543070f4fd3e96f3183cae9ee2ccbe843409ee58 core/src/test/scala/unit/kafka/consumer/ZookeeperConsumerConnectorTest.scala a17e8532c44aadf84b8da3a57bcc797a848b5020 Diff: https://reviews.apache.org/r/31706/diff/ Testing --- Thanks, Jiangjie Qin
Re: Review Request 31650: Drag Coordinator and FetchManager out of KafkaConsumer, fix a bunch of consumer test issues
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/31650/ --- (Updated March 5, 2015, 10:57 p.m.) Review request for kafka. Bugs: KAFKA-1910 https://issues.apache.org/jira/browse/KAFKA-1910 Repository: kafka Description (updated) --- See comments in KAFKA-1910; Updated RB includes unit test for Coordinator / FetchManager / Heartbeat and a couple changes on MemoryRecords and test utils. Diffs - clients/src/main/java/org/apache/kafka/clients/CommonClientConfigs.java 06fcfe62cc1fe76f58540221698ef076fe150e96 clients/src/main/java/org/apache/kafka/clients/KafkaClient.java 8a3e55aaff7d8c26e56a8407166a4176c1da2644 clients/src/main/java/org/apache/kafka/clients/NetworkClient.java a7fa4a9dfbcfbc4d9e9259630253cbcded158064 clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerConfig.java 5fb21001abd77cac839bd724afa04e377a3e82aa clients/src/main/java/org/apache/kafka/clients/consumer/KafkaConsumer.java 67ceb754a52c07143c69b053fe128b9e24060b99 clients/src/main/java/org/apache/kafka/clients/consumer/internals/Coordinator.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/internals/FetchManager.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/internals/Heartbeat.java ee0751e4949120d114202c2299d49612a89b9d97 clients/src/main/java/org/apache/kafka/clients/consumer/internals/SubscriptionState.java d41d3068c11d4b5c640467dc0ae1b7c20a8d128c clients/src/main/java/org/apache/kafka/clients/producer/KafkaProducer.java 7397e565fd865214529ffccadd4222d835ac8110 clients/src/main/java/org/apache/kafka/clients/producer/ProducerConfig.java 122375c473bf73caf05299b9f5174c6b226ca863 clients/src/main/java/org/apache/kafka/clients/producer/internals/Sender.java ed9c63a6679e3aaf83d19fde19268553a4c107c2 clients/src/main/java/org/apache/kafka/common/network/Selector.java 6baad9366a1975dbaba1786da91efeaa38533319 clients/src/main/java/org/apache/kafka/common/protocol/Errors.java ad2171f5417c93194f5f234bdc7fdd0b8d59a8a8 clients/src/main/java/org/apache/kafka/common/record/MemoryRecords.java 083e7a39249ab56a73a014b106876244d619f189 clients/src/main/java/org/apache/kafka/common/requests/FetchResponse.java e67c4c8332cb1dd3d9cde5de687df7760045dfe6 clients/src/main/java/org/apache/kafka/common/requests/HeartbeatResponse.java 0057496228feeeccbc0c009a42f5268fa2cb8611 clients/src/main/java/org/apache/kafka/common/requests/JoinGroupRequest.java 8c50e9be534c61ecf56106bf2b68cf678ea50d66 clients/src/main/java/org/apache/kafka/common/requests/JoinGroupResponse.java 52b1803d8b558c1eeb978ba8821496c7d3c20a6b clients/src/main/java/org/apache/kafka/common/requests/ListOffsetResponse.java cfac47a4a05dc8a535595542d93e55237b7d1e93 clients/src/main/java/org/apache/kafka/common/requests/MetadataResponse.java 90f31413d7d80a06c0af359009cc271aa0c67be3 clients/src/main/java/org/apache/kafka/common/requests/OffsetCommitResponse.java 4d3b9ececee4b4c0b50ba99da2ddbbb15f9cc08d clients/src/main/java/org/apache/kafka/common/requests/OffsetFetchResponse.java edbed5880dc44fc178737a5e298c106a00f38443 clients/src/main/java/org/apache/kafka/common/requests/ProduceResponse.java a00dcdf15d1c7bac7228be140647bd7d849deb9b clients/src/test/java/org/apache/kafka/clients/MockClient.java 8f1a7a625e4eeafa44bbf9e5cff987de86c949be clients/src/test/java/org/apache/kafka/clients/consumer/internals/CoordinatorTest.java PRE-CREATION clients/src/test/java/org/apache/kafka/clients/consumer/internals/FetchManagerTest.java PRE-CREATION clients/src/test/java/org/apache/kafka/clients/consumer/internals/HeartbeatTest.java PRE-CREATION clients/src/test/java/org/apache/kafka/clients/consumer/internals/SubscriptionStateTest.java 090087a319e2697d3a0653ca947d2cfa6d53f6c2 clients/src/test/java/org/apache/kafka/clients/producer/internals/RecordAccumulatorTest.java c1bc40648479d4c2ae4ac52f40dadc070a6bcf6f clients/src/test/java/org/apache/kafka/clients/producer/internals/SenderTest.java ea56c997cb90d5bac8e3706dfc1eaae5b1555ccb clients/src/test/java/org/apache/kafka/common/record/MemoryRecordsTest.java e343327faf115a901657ec6da8e0c5b8bbf0b76a core/src/main/scala/kafka/common/ErrorMapping.scala eedc2f5f21dd8755fba891998456351622e17047 core/src/main/scala/kafka/common/NoOffsetsCommittedException.scala PRE-CREATION core/src/main/scala/kafka/common/OffsetMetadataAndError.scala 4cabffeacea09a49913505db19a96a55d58c0909 core/src/main/scala/kafka/coordinator/ConsumerCoordinator.scala 21790a5059ee00d6610be6f0389445327b88db1d core/src/main/scala/kafka/coordinator/ConsumerRegistry.scala b65c04d0a5d53bf92299d5f67f112be3da3bf77d core/src/main/scala/kafka/coordinator/DelayedHeartbeat.scala b1248e95d8a648b461f604c154879cc95dc7b1cb
[jira] [Commented] (KAFKA-1646) Improve consumer read performance for Windows
[ https://issues.apache.org/jira/browse/KAFKA-1646?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14349621#comment-14349621 ] Honghai Chen commented on KAFKA-1646: - Or do you prefer the second option: Add one more column to file recovery-point-offset-checkpoint, currently it only record offset, like below: 0 2 mvlogs 1 100 mvlogs 0 200 Change to below: 0 2 mvlogs 1 100 8000 mvlogs 0 200 16000 8000 is the start position of the data file for message with offset 100 . And 16000 is start position of the data file for message with offset 200. Take first one as example, what we need do are: 1, keep offset and position consistent and regularly write to file recovery-point-offset-checkpoint, 2, when in clean shutdown, truncate the file to the size. 3, when start, if the os is windows, add one new segment. But this change is big, since so many places are using variable recoveryPoint. Which one do you recommend? Really appreciate for your guide. [~jkreps][~nehanarkhede][~junrao] Improve consumer read performance for Windows - Key: KAFKA-1646 URL: https://issues.apache.org/jira/browse/KAFKA-1646 Project: Kafka Issue Type: Improvement Components: log Affects Versions: 0.8.1.1 Environment: Windows Reporter: xueqiang wang Assignee: xueqiang wang Labels: newbie, patch Attachments: Improve consumer read performance for Windows.patch, KAFKA-1646-truncate-off-trailing-zeros-on-broker-restart-if-bro.patch, KAFKA-1646_20141216_163008.patch This patch is for Window platform only. In Windows platform, if there are more than one replicas writing to disk, the segment log files will not be consistent in disk and then consumer reading performance will be dropped down greatly. This fix allocates more disk spaces when rolling a new segment, and then it will improve the consumer reading performance in NTFS file system. This patch doesn't affect file allocation of other filesystems, for it only adds statements like 'if(Os.iswindow)' or adds methods used on Windows. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Updated] (KAFKA-1910) Refactor KafkaConsumer
[ https://issues.apache.org/jira/browse/KAFKA-1910?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Guozhang Wang updated KAFKA-1910: - Attachment: KAFKA-1910_2015-03-05_14:55:33.patch Refactor KafkaConsumer -- Key: KAFKA-1910 URL: https://issues.apache.org/jira/browse/KAFKA-1910 Project: Kafka Issue Type: Sub-task Components: consumer Reporter: Guozhang Wang Assignee: Guozhang Wang Attachments: KAFKA-1910.patch, KAFKA-1910_2015-03-05_14:55:33.patch KafkaConsumer now contains all the logic on the consumer side, making it a very huge class file, better re-factoring it to have multiple layers on top of KafkaClient. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
Re: Review Request 31650: Drag Coordinator and FetchManager out of KafkaConsumer, fix a bunch of consumer test issues
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/31650/#review75411 --- clients/src/main/java/org/apache/kafka/clients/CommonClientConfigs.java https://reviews.apache.org/r/31650/#comment122448 Not sure if I get this--isn't the serializer specific to the producer and deserializer specific to the consumer? clients/src/main/java/org/apache/kafka/clients/consumer/KafkaConsumer.java https://reviews.apache.org/r/31650/#comment122455 Do we need to inline all this in the main poll loop? Is it possible to break this stuff out into a seperate method just to be able to follow the logic. I think this block is basically something like reassignPartitions() ? clients/src/main/java/org/apache/kafka/clients/consumer/internals/Coordinator.java https://reviews.apache.org/r/31650/#comment122450 Can you move the inner classes to the bottom as we do everywhere else? clients/src/main/java/org/apache/kafka/clients/consumer/internals/FetchManager.java https://reviews.apache.org/r/31650/#comment122451 Manager is always such a generic name, what do you think about calling this the Fetcher to match the Sender that the Producer has? I think this refactoring looks reasonable. I'll take a deeper look but I think since the goal is to move quick on this stuff, you should consider just checking in now and we can do any additional items as follow-ups. - Jay Kreps On March 5, 2015, 10:57 p.m., Guozhang Wang wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/31650/ --- (Updated March 5, 2015, 10:57 p.m.) Review request for kafka. Bugs: KAFKA-1910 https://issues.apache.org/jira/browse/KAFKA-1910 Repository: kafka Description --- See comments in KAFKA-1910; Updated RB includes unit test for Coordinator / FetchManager / Heartbeat and a couple changes on MemoryRecords and test utils. Diffs - clients/src/main/java/org/apache/kafka/clients/CommonClientConfigs.java 06fcfe62cc1fe76f58540221698ef076fe150e96 clients/src/main/java/org/apache/kafka/clients/KafkaClient.java 8a3e55aaff7d8c26e56a8407166a4176c1da2644 clients/src/main/java/org/apache/kafka/clients/NetworkClient.java a7fa4a9dfbcfbc4d9e9259630253cbcded158064 clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerConfig.java 5fb21001abd77cac839bd724afa04e377a3e82aa clients/src/main/java/org/apache/kafka/clients/consumer/KafkaConsumer.java 67ceb754a52c07143c69b053fe128b9e24060b99 clients/src/main/java/org/apache/kafka/clients/consumer/internals/Coordinator.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/internals/FetchManager.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/internals/Heartbeat.java ee0751e4949120d114202c2299d49612a89b9d97 clients/src/main/java/org/apache/kafka/clients/consumer/internals/SubscriptionState.java d41d3068c11d4b5c640467dc0ae1b7c20a8d128c clients/src/main/java/org/apache/kafka/clients/producer/KafkaProducer.java 7397e565fd865214529ffccadd4222d835ac8110 clients/src/main/java/org/apache/kafka/clients/producer/ProducerConfig.java 122375c473bf73caf05299b9f5174c6b226ca863 clients/src/main/java/org/apache/kafka/clients/producer/internals/Sender.java ed9c63a6679e3aaf83d19fde19268553a4c107c2 clients/src/main/java/org/apache/kafka/common/network/Selector.java 6baad9366a1975dbaba1786da91efeaa38533319 clients/src/main/java/org/apache/kafka/common/protocol/Errors.java ad2171f5417c93194f5f234bdc7fdd0b8d59a8a8 clients/src/main/java/org/apache/kafka/common/record/MemoryRecords.java 083e7a39249ab56a73a014b106876244d619f189 clients/src/main/java/org/apache/kafka/common/requests/FetchResponse.java e67c4c8332cb1dd3d9cde5de687df7760045dfe6 clients/src/main/java/org/apache/kafka/common/requests/HeartbeatResponse.java 0057496228feeeccbc0c009a42f5268fa2cb8611 clients/src/main/java/org/apache/kafka/common/requests/JoinGroupRequest.java 8c50e9be534c61ecf56106bf2b68cf678ea50d66 clients/src/main/java/org/apache/kafka/common/requests/JoinGroupResponse.java 52b1803d8b558c1eeb978ba8821496c7d3c20a6b clients/src/main/java/org/apache/kafka/common/requests/ListOffsetResponse.java cfac47a4a05dc8a535595542d93e55237b7d1e93 clients/src/main/java/org/apache/kafka/common/requests/MetadataResponse.java 90f31413d7d80a06c0af359009cc271aa0c67be3 clients/src/main/java/org/apache/kafka/common/requests/OffsetCommitResponse.java 4d3b9ececee4b4c0b50ba99da2ddbbb15f9cc08d clients/src/main/java/org/apache/kafka/common/requests/OffsetFetchResponse.java
Re: Review Request 31650: Drag Coordinator and FetchManager out of KafkaConsumer, fix a bunch of consumer test issues
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/31650/#review75355 --- clients/src/main/java/org/apache/kafka/clients/CommonClientConfigs.java https://reviews.apache.org/r/31650/#comment122365 Do key/value serializer/deserializer configs actually belong here? CommonClientConfigs is where shared configs for producers and consumers live. A producer is only interested in key/value serializer config and consumer is only interested in key/value deserializer config. If you are trying to move the addSerializerToConfig and addDeserializerToConfig out of the KafkaProducer and KafkaConsumer, maybe the ProducerConfig and ConsumerConfig would be a suitable place? clients/src/main/java/org/apache/kafka/clients/consumer/internals/Coordinator.java https://reviews.apache.org/r/31650/#comment122377 This class name was very misleading to me. It should be renamed such that it's clear that it's used by clients to interact with coordinators, and is not the actual coordinator. Maybe CoordinatorClient, CoordinatorManager, or something else that makes this clear. clients/src/main/java/org/apache/kafka/clients/consumer/internals/Coordinator.java https://reviews.apache.org/r/31650/#comment122397 This is really minor, but are longs necessary for these time parameters? Integer.MAX_VALUE translates to a little over 24 days. clients/src/main/java/org/apache/kafka/clients/consumer/internals/Coordinator.java https://reviews.apache.org/r/31650/#comment122396 This is marking the receivedResponse as the time the request was sent rather than the time we received the response. clients/src/main/java/org/apache/kafka/clients/consumer/internals/Coordinator.java https://reviews.apache.org/r/31650/#comment122421 I think this is simpler as: ```java boolean done = false; while (!done) { } ``` clients/src/main/java/org/apache/kafka/clients/consumer/internals/Coordinator.java https://reviews.apache.org/r/31650/#comment122384 unless you plan on adding a condition to the loop, this can just be a while(true) loop. clients/src/main/java/org/apache/kafka/clients/consumer/internals/Coordinator.java https://reviews.apache.org/r/31650/#comment122422 `if (responses.isEmpty())` clients/src/main/java/org/apache/kafka/common/protocol/Errors.java https://reviews.apache.org/r/31650/#comment122360 Using the term consumer implies that generation ids are associated with a consumer, while they're really associated with a group. Maybe just call this ILLEGAL_GENERATION as stated in the wiki? https://cwiki.apache.org/confluence/display/KAFKA/Kafka+0.9+Consumer+Rewrite+Design#Kafka0.9ConsumerRewriteDesign-Groupmanagementprotocol core/src/main/scala/kafka/coordinator/GroupRegistry.scala https://reviews.apache.org/r/31650/#comment122338 var to val for both of these. core/src/main/scala/kafka/coordinator/GroupRegistry.scala https://reviews.apache.org/r/31650/#comment122339 the toString is not needed. core/src/test/scala/integration/kafka/api/ConsumerTest.scala https://reviews.apache.org/r/31650/#comment122359 It would be nice to see unit tests specifically covering FetchManager. - Onur Karaman On March 3, 2015, 12:46 a.m., Guozhang Wang wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/31650/ --- (Updated March 3, 2015, 12:46 a.m.) Review request for kafka. Bugs: KAFKA-1910 https://issues.apache.org/jira/browse/KAFKA-1910 Repository: kafka Description --- See comments in KAFKA-1910 Diffs - clients/src/main/java/org/apache/kafka/clients/CommonClientConfigs.java 06fcfe62cc1fe76f58540221698ef076fe150e96 clients/src/main/java/org/apache/kafka/clients/KafkaClient.java 8a3e55aaff7d8c26e56a8407166a4176c1da2644 clients/src/main/java/org/apache/kafka/clients/NetworkClient.java a7fa4a9dfbcfbc4d9e9259630253cbcded158064 clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerConfig.java 5fb21001abd77cac839bd724afa04e377a3e82aa clients/src/main/java/org/apache/kafka/clients/consumer/KafkaConsumer.java 67ceb754a52c07143c69b053fe128b9e24060b99 clients/src/main/java/org/apache/kafka/clients/consumer/internals/Coordinator.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/internals/FetchManager.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/internals/Heartbeat.java ee0751e4949120d114202c2299d49612a89b9d97 clients/src/main/java/org/apache/kafka/clients/consumer/internals/SubscriptionState.java d41d3068c11d4b5c640467dc0ae1b7c20a8d128c
Re: Review Request 31650: Drag Coordinator and FetchManager out of KafkaConsumer, fix a bunch of consumer test issues
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/31650/#review75419 --- Ship it! Ship It! - Jay Kreps On March 5, 2015, 10:57 p.m., Guozhang Wang wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/31650/ --- (Updated March 5, 2015, 10:57 p.m.) Review request for kafka. Bugs: KAFKA-1910 https://issues.apache.org/jira/browse/KAFKA-1910 Repository: kafka Description --- See comments in KAFKA-1910; Updated RB includes unit test for Coordinator / FetchManager / Heartbeat and a couple changes on MemoryRecords and test utils. Diffs - clients/src/main/java/org/apache/kafka/clients/CommonClientConfigs.java 06fcfe62cc1fe76f58540221698ef076fe150e96 clients/src/main/java/org/apache/kafka/clients/KafkaClient.java 8a3e55aaff7d8c26e56a8407166a4176c1da2644 clients/src/main/java/org/apache/kafka/clients/NetworkClient.java a7fa4a9dfbcfbc4d9e9259630253cbcded158064 clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerConfig.java 5fb21001abd77cac839bd724afa04e377a3e82aa clients/src/main/java/org/apache/kafka/clients/consumer/KafkaConsumer.java 67ceb754a52c07143c69b053fe128b9e24060b99 clients/src/main/java/org/apache/kafka/clients/consumer/internals/Coordinator.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/internals/FetchManager.java PRE-CREATION clients/src/main/java/org/apache/kafka/clients/consumer/internals/Heartbeat.java ee0751e4949120d114202c2299d49612a89b9d97 clients/src/main/java/org/apache/kafka/clients/consumer/internals/SubscriptionState.java d41d3068c11d4b5c640467dc0ae1b7c20a8d128c clients/src/main/java/org/apache/kafka/clients/producer/KafkaProducer.java 7397e565fd865214529ffccadd4222d835ac8110 clients/src/main/java/org/apache/kafka/clients/producer/ProducerConfig.java 122375c473bf73caf05299b9f5174c6b226ca863 clients/src/main/java/org/apache/kafka/clients/producer/internals/Sender.java ed9c63a6679e3aaf83d19fde19268553a4c107c2 clients/src/main/java/org/apache/kafka/common/network/Selector.java 6baad9366a1975dbaba1786da91efeaa38533319 clients/src/main/java/org/apache/kafka/common/protocol/Errors.java ad2171f5417c93194f5f234bdc7fdd0b8d59a8a8 clients/src/main/java/org/apache/kafka/common/record/MemoryRecords.java 083e7a39249ab56a73a014b106876244d619f189 clients/src/main/java/org/apache/kafka/common/requests/FetchResponse.java e67c4c8332cb1dd3d9cde5de687df7760045dfe6 clients/src/main/java/org/apache/kafka/common/requests/HeartbeatResponse.java 0057496228feeeccbc0c009a42f5268fa2cb8611 clients/src/main/java/org/apache/kafka/common/requests/JoinGroupRequest.java 8c50e9be534c61ecf56106bf2b68cf678ea50d66 clients/src/main/java/org/apache/kafka/common/requests/JoinGroupResponse.java 52b1803d8b558c1eeb978ba8821496c7d3c20a6b clients/src/main/java/org/apache/kafka/common/requests/ListOffsetResponse.java cfac47a4a05dc8a535595542d93e55237b7d1e93 clients/src/main/java/org/apache/kafka/common/requests/MetadataResponse.java 90f31413d7d80a06c0af359009cc271aa0c67be3 clients/src/main/java/org/apache/kafka/common/requests/OffsetCommitResponse.java 4d3b9ececee4b4c0b50ba99da2ddbbb15f9cc08d clients/src/main/java/org/apache/kafka/common/requests/OffsetFetchResponse.java edbed5880dc44fc178737a5e298c106a00f38443 clients/src/main/java/org/apache/kafka/common/requests/ProduceResponse.java a00dcdf15d1c7bac7228be140647bd7d849deb9b clients/src/test/java/org/apache/kafka/clients/MockClient.java 8f1a7a625e4eeafa44bbf9e5cff987de86c949be clients/src/test/java/org/apache/kafka/clients/consumer/internals/CoordinatorTest.java PRE-CREATION clients/src/test/java/org/apache/kafka/clients/consumer/internals/FetchManagerTest.java PRE-CREATION clients/src/test/java/org/apache/kafka/clients/consumer/internals/HeartbeatTest.java PRE-CREATION clients/src/test/java/org/apache/kafka/clients/consumer/internals/SubscriptionStateTest.java 090087a319e2697d3a0653ca947d2cfa6d53f6c2 clients/src/test/java/org/apache/kafka/clients/producer/internals/RecordAccumulatorTest.java c1bc40648479d4c2ae4ac52f40dadc070a6bcf6f clients/src/test/java/org/apache/kafka/clients/producer/internals/SenderTest.java ea56c997cb90d5bac8e3706dfc1eaae5b1555ccb clients/src/test/java/org/apache/kafka/common/record/MemoryRecordsTest.java e343327faf115a901657ec6da8e0c5b8bbf0b76a core/src/main/scala/kafka/common/ErrorMapping.scala eedc2f5f21dd8755fba891998456351622e17047 core/src/main/scala/kafka/common/NoOffsetsCommittedException.scala PRE-CREATION
[jira] [Comment Edited] (KAFKA-1646) Improve consumer read performance for Windows
[ https://issues.apache.org/jira/browse/KAFKA-1646?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14349621#comment-14349621 ] Honghai Chen edited comment on KAFKA-1646 at 3/5/15 11:35 PM: -- Or do you prefer the second option: Add one more column to file recovery-point-offset-checkpoint, currently it only record offset, like below: 0 2 mvlogs 1 100 mvlogs 0 200 Change to below by add one column recoverposition 0 2 mvlogs 1 100 8000 mvlogs 0 200 16000 8000 is the start position of the data file for message with offset 100 . And 16000 is start position of the data file for message with offset 200. Take first one as example, what we need do are: 1, keep offset and position consistent and regularly write to file recovery-point-offset-checkpoint, 2, when in clean shutdown, truncate the file to the recoverposition. 3, when start, find the log segment related with the recover point, truncate the file to the recoverposition 4, when start, if the os is windows, add one new segment. But this change is big, since so many places are using variable recoveryPoint. Which one do you recommend? Really appreciate for your guide. [~jkreps][~nehanarkhede][~junrao] was (Author: waldenchen): Or do you prefer the second option: Add one more column to file recovery-point-offset-checkpoint, currently it only record offset, like below: 0 2 mvlogs 1 100 mvlogs 0 200 Change to below: 0 2 mvlogs 1 100 8000 mvlogs 0 200 16000 8000 is the start position of the data file for message with offset 100 . And 16000 is start position of the data file for message with offset 200. Take first one as example, what we need do are: 1, keep offset and position consistent and regularly write to file recovery-point-offset-checkpoint, 2, when in clean shutdown, truncate the file to the size. 3, when start, if the os is windows, add one new segment. But this change is big, since so many places are using variable recoveryPoint. Which one do you recommend? Really appreciate for your guide. [~jkreps][~nehanarkhede][~junrao] Improve consumer read performance for Windows - Key: KAFKA-1646 URL: https://issues.apache.org/jira/browse/KAFKA-1646 Project: Kafka Issue Type: Improvement Components: log Affects Versions: 0.8.1.1 Environment: Windows Reporter: xueqiang wang Assignee: xueqiang wang Labels: newbie, patch Attachments: Improve consumer read performance for Windows.patch, KAFKA-1646-truncate-off-trailing-zeros-on-broker-restart-if-bro.patch, KAFKA-1646_20141216_163008.patch This patch is for Window platform only. In Windows platform, if there are more than one replicas writing to disk, the segment log files will not be consistent in disk and then consumer reading performance will be dropped down greatly. This fix allocates more disk spaces when rolling a new segment, and then it will improve the consumer reading performance in NTFS file system. This patch doesn't affect file allocation of other filesystems, for it only adds statements like 'if(Os.iswindow)' or adds methods used on Windows. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
Re: Review Request 31627: Patch for KAFKA-1884
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/31627/#review75343 --- Ship it! Ship It! - Sriharsha Chintalapani On March 2, 2015, 3:57 p.m., Manikumar Reddy O wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/31627/ --- (Updated March 2, 2015, 3:57 p.m.) Review request for kafka. Bugs: KAFKA-1884 https://issues.apache.org/jira/browse/KAFKA-1884 Repository: kafka Description --- Added logs to print metadata response errors Diffs - clients/src/main/java/org/apache/kafka/clients/NetworkClient.java a7fa4a9dfbcfbc4d9e9259630253cbcded158064 Diff: https://reviews.apache.org/r/31627/diff/ Testing --- Thanks, Manikumar Reddy O
Re: [kafka-clients] Re: [VOTE] 0.8.2.1 Candidate 2
+1 unit tests and ran tests on 3-node cluster. On Wed, Mar 4, 2015, at 09:01 PM, Neha Narkhede wrote: +1. Verified quick start, unit tests. On Tue, Mar 3, 2015 at 12:09 PM, Joe Stein joe.st...@stealth.ly wrote: Ok, lets fix the transient test failure on trunk agreed not a blocker. +1 quick start passed, verified artifacts, updates in scala https://github.com/stealthly/scala-kafka/tree/0.8.2.1 and go https://github.com/stealthly/go_kafka_client/tree/0.8.2.1 look good ~ Joe Stein - - - - - - - - - - - - - - - - - http://www.stealth.ly - - - - - - - - - - - - - - - - - On Tue, Mar 3, 2015 at 12:30 PM, Jun Rao j...@confluent.io wrote: Hi, Joe, Yes, that unit test does have transient failures from time to time. The issue seems to be with the unit test itself and not the actual code. So, this is not a blocker for 0.8.2.1 release. I think we can just fix it in trunk. Thanks, Jun On Tue, Mar 3, 2015 at 9:08 AM, Joe Stein joe.st...@stealth.ly wrote: Jun, I have most everything looks good except I keep getting test failures from wget https://people.apache.org/~junrao/kafka-0.8.2.1-candidate2/kafka-0.8.2.1-src.tgz tar -xvf kafka-0.8.2.1-src.tgz cd kafka-0.8.2.1-src gradle ./gradlew test kafka.api.ProducerFailureHandlingTest testNotEnoughReplicasAfterBrokerShutdown FAILED org.scalatest.junit.JUnitTestFailedError: Expected NotEnoughReplicasException when producing to topic with fewer brokers than min.insync.replicas at org.scalatest.junit.AssertionsForJUnit$class.newAssertionFailedException(AssertionsForJUnit.scala:101) at org.scalatest.junit.JUnit3Suite.newAssertionFailedException(JUnit3Suite.scala:149) at org.scalatest.Assertions$class.fail(Assertions.scala:711) at org.scalatest.junit.JUnit3Suite.fail(JUnit3Suite.scala:149) at kafka.api.ProducerFailureHandlingTest.testNotEnoughReplicasAfterBrokerShutdown(ProducerFailureHandlingTest.scala:355) This happens to me all the time on a few different machines. ~ Joe Stein - - - - - - - - - - - - - - - - - http://www.stealth.ly - - - - - - - - - - - - - - - - - On Mon, Mar 2, 2015 at 7:36 PM, Jun Rao j...@confluent.io wrote: +1 from me. Verified quickstart and unit tests. Thanks, Jun On Thu, Feb 26, 2015 at 2:59 PM, Jun Rao j...@confluent.io wrote: This is the second candidate for release of Apache Kafka 0.8.2.1. This fixes 4 critical issue in 0.8.2.0. Release Notes for the 0.8.2.1 release https://people.apache.org/~junrao/kafka-0.8.2.1-candidate2/RELEASE_NOTES.html *** Please download, test and vote by Monday, Mar 2, 3pm PT Kafka's KEYS file containing PGP keys we use to sign the release: http://kafka.apache.org/KEYS in addition to the md5, sha1 and sha2 (SHA256) checksum. * Release artifacts to be voted upon (source and binary): https://people.apache.org/~junrao/kafka-0.8.2.1-candidate2/ * Maven artifacts to be voted upon prior to release: https://repository.apache.org/content/groups/staging/ * scala-doc https://people.apache.org/~junrao/kafka-0.8.2.1-candidate2/scaladoc/ * java-doc https://people.apache.org/~junrao/kafka-0.8.2.1-candidate2/javadoc/ * The tag to be voted upon (off the 0.8.2 branch) is the 0.8.2.1 tag https://git-wip-us.apache.org/repos/asf?p=kafka.git;a=tag;h=bd1bfb63ec73c10d08432ac893a23f28281ea021 (git commit ee1267b127f3081db491fa1bf9a287084c324e36) /*** Thanks, Jun -- You received this message because you are subscribed to the Google Groups kafka-clients group. To unsubscribe from this group and stop receiving emails from it, send an email to kafka-clients+unsubscr...@googlegroups.com. To post to this group, send email to kafka-clie...@googlegroups.com. Visit this group at http://groups.google.com/group/kafka-clients. To view this discussion on the web visit https://groups.google.com/d/msgid/kafka-clients/CAFc58G_5FuLKx3_kM4PCgqQL8d%2B4sqE0o-%2BXfu3FJicAgn5KPw%40mail.gmail.com https://groups.google.com/d/msgid/kafka-clients/CAFc58G_5FuLKx3_kM4PCgqQL8d%2B4sqE0o-%2BXfu3FJicAgn5KPw%40mail.gmail.com?utm_medium=emailutm_source=footer . For more options, visit https://groups.google.com/d/optout. -- Thanks, Neha
[jira] [Updated] (KAFKA-1884) Print metadata response errors
[ https://issues.apache.org/jira/browse/KAFKA-1884?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Manikumar Reddy updated KAFKA-1884: --- Description: Print metadata response errors. producer logs: DEBUG [2015-01-20 12:46:13,406] NetworkClient: maybeUpdateMetadata(): Trying to send metadata request to node -1 DEBUG [2015-01-20 12:46:13,406] NetworkClient: maybeUpdateMetadata(): Sending metadata request ClientRequest(expectResponse=true, payload=null, request=RequestSend(header={api_key=3,api_version=0,correlation_id=50845,client_id=my-producer}, body={topics=[TOPIC=]})) to node -1 TRACE [2015-01-20 12:46:13,416] NetworkClient: handleMetadataResponse(): Ignoring empty metadata response with correlation id 50845. DEBUG [2015-01-20 12:46:13,417] NetworkClient: maybeUpdateMetadata(): Trying to send metadata request to node -1 DEBUG [2015-01-20 12:46:13,417] NetworkClient: maybeUpdateMetadata(): Sending metadata request ClientRequest(expectResponse=true, payload=null, request=RequestSend(header={api_key=3,api_version=0,correlation_id=50846,client_id=my-producer}, body={topics=[TOPIC=]})) to node -1 TRACE [2015-01-20 12:46:13,417] NetworkClient: handleMetadataResponse(): Ignoring empty metadata response with correlation id 50846. DEBUG [2015-01-20 12:46:13,417] NetworkClient: maybeUpdateMetadata(): Trying to send metadata request to node -1 DEBUG [2015-01-20 12:46:13,418] NetworkClient: maybeUpdateMetadata(): Sending metadata request ClientRequest(expectResponse=true, payload=null, request=RequestSend(header={api_key=3,api_version=0,correlation_id=50847,client_id=my-producer}, body={topics=[TOPIC=]})) to node -1 TRACE [2015-01-20 12:46:13,418] NetworkClient: handleMetadataResponse(): Ignoring empty metadata response with correlation id 50847. Broker logs: [2015-01-20 12:46:14,074] ERROR [KafkaApi-0] error when handling request Name: TopicMetadataRequest; Version: 0; CorrelationId: 51020; ClientId: my-producer; Topics: TOPIC= (kafka.server.KafkaApis) kafka.common.InvalidTopicException: topic name TOPIC= is illegal, contains a character other than ASCII alphanumerics, '.', '_' and '-' at kafka.common.Topic$.validate(Topic.scala:42) at kafka.admin.AdminUtils$.createOrUpdateTopicPartitionAssignmentPathInZK(AdminUtils.scala:186) at kafka.admin.AdminUtils$.createTopic(AdminUtils.scala:177) at kafka.server.KafkaApis$$anonfun$5.apply(KafkaApis.scala:367) at kafka.server.KafkaApis$$anonfun$5.apply(KafkaApis.scala:350) at scala.collection.TraversableLike$$anonfun$map$1.apply(TraversableLike.scala:244) at scala.collection.TraversableLike$$anonfun$map$1.apply(TraversableLike.scala:244) at scala.collection.immutable.Set$Set1.foreach(Set.scala:74) at scala.collection.TraversableLike$class.map(TraversableLike.scala:244) at scala.collection.AbstractSet.scala$collection$SetLike$$super$map(Set.scala:47) at scala.collection.SetLike$class.map(SetLike.scala:93) at scala.collection.AbstractSet.map(Set.scala:47) at kafka.server.KafkaApis.getTopicMetadata(KafkaApis.scala:350) at kafka.server.KafkaApis.handleTopicMetadataRequest(KafkaApis.scala:389) at kafka.server.KafkaApis.handle(KafkaApis.scala:57) at kafka.server.KafkaRequestHandler.run(KafkaRequestHandler.scala:59) at java.lang.Thread.run(Thread.java:722) was: New producer blocks forever for invalid topics names producer logs: DEBUG [2015-01-20 12:46:13,406] NetworkClient: maybeUpdateMetadata(): Trying to send metadata request to node -1 DEBUG [2015-01-20 12:46:13,406] NetworkClient: maybeUpdateMetadata(): Sending metadata request ClientRequest(expectResponse=true, payload=null, request=RequestSend(header={api_key=3,api_version=0,correlation_id=50845,client_id=my-producer}, body={topics=[TOPIC=]})) to node -1 TRACE [2015-01-20 12:46:13,416] NetworkClient: handleMetadataResponse(): Ignoring empty metadata response with correlation id 50845. DEBUG [2015-01-20 12:46:13,417] NetworkClient: maybeUpdateMetadata(): Trying to send metadata request to node -1 DEBUG [2015-01-20 12:46:13,417] NetworkClient: maybeUpdateMetadata(): Sending metadata request ClientRequest(expectResponse=true, payload=null, request=RequestSend(header={api_key=3,api_version=0,correlation_id=50846,client_id=my-producer}, body={topics=[TOPIC=]})) to node -1 TRACE [2015-01-20 12:46:13,417] NetworkClient: handleMetadataResponse(): Ignoring empty metadata response with correlation id 50846. DEBUG [2015-01-20 12:46:13,417] NetworkClient: maybeUpdateMetadata(): Trying to send metadata request to node -1 DEBUG [2015-01-20 12:46:13,418] NetworkClient: maybeUpdateMetadata(): Sending metadata request ClientRequest(expectResponse=true, payload=null, request=RequestSend(header={api_key=3,api_version=0,correlation_id=50847,client_id=my-producer}, body={topics=[TOPIC=]})) to node -1
[jira] [Commented] (KAFKA-1754) KOYA - Kafka on YARN
[ https://issues.apache.org/jira/browse/KAFKA-1754?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14349070#comment-14349070 ] Ravi Prakash commented on KAFKA-1754: - Thanks for the talk yesterday Thomas discussing Kafka on YARN via Slider! Are there still plans to create a new Kafka AM or is Slider the way forward? KOYA - Kafka on YARN Key: KAFKA-1754 URL: https://issues.apache.org/jira/browse/KAFKA-1754 Project: Kafka Issue Type: New Feature Reporter: Thomas Weise Attachments: DT-KOYA-Proposal- JIRA.pdf YARN (Hadoop 2.x) has enabled clusters to be used for a variety of workloads, emerging as distributed operating system for big data applications. Initiatives are on the way to bring long running services under the YARN umbrella, leveraging it for centralized resource management and operations ([YARN-896] and examples such as HBase, Accumulo or Memcached through Slider). This JIRA is to propose KOYA (Kafka On Yarn), a YARN application master to launch and manage Kafka clusters running on YARN. Brokers will use resources allocated through YARN with support for recovery, monitoring etc. Please see attached for more details. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Resolved] (KAFKA-1998) Partitions Missing From MetadataResponse
[ https://issues.apache.org/jira/browse/KAFKA-1998?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Guozhang Wang resolved KAFKA-1998. -- Resolution: Not a Problem Partitions Missing From MetadataResponse Key: KAFKA-1998 URL: https://issues.apache.org/jira/browse/KAFKA-1998 Project: Kafka Issue Type: Bug Components: core Affects Versions: 0.8.2.0 Reporter: Evan Huus Assignee: Mayuresh Gharat It is known behaviour that when a partition is entirely offline (it has no leader because all of its replicas are down) then that partition will not be included in the metadata returned by other brokers. For example, if topic foo has 3 partitions, but all replicas of partition 3 are offline, then requesting metadata for foo will only return information about partitions 1 and 2. This means that there is no way to reliably determine the number of partitions for a topic via kafka's metadata API; if I receive information on partitions 1 and 2, I don't know if partition 3 is offline or if it is simply that there are only two partitions total. (You can presumably still ask zookeeper directly, but that is a work-around). This ambiguity, in turn, can lead to a consistency problem with the default partitioner, since that effectively implements `hash(key) mod #partitions`. If a partition goes offline and is removed from the metadata response, then the number of partitions the producer knows about will change (on its next metadata refresh) and the mapping from keys to partitions will also change. Instead of distributing messages among (for example) 3 partitions, and failing to produce to the offline partition, it will distribute *all* messages among the two online partitions. This results in messages being sent to the wrong partition. Since kafka already returns partitions with error messages in many cases (e.g. `LeaderNotAvailable`) I think it makes much more sense and fixes the above partition problem if it would simply return offline partitions as well with the appropriate error (whether that is `LeaderNotAvailable` or it would be better to add an additional error is up to you). CC [~guozhang] (This issue was originally described/discussed on the kafka-users mailing list, in the thread involving https://mail-archives.apache.org/mod_mbox/kafka-users/201503.mbox/%3CCAA4pprAZvp2XhdNmy0%2BqVZ1UVdVxmUfz3DDArhGbwP-iiH%2BGyg%40mail.gmail.com%3E) If there are any questions I am happy to clarify, I realize the scenario is somewhat complex. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Commented] (KAFKA-1998) Partitions Missing From MetadataResponse
[ https://issues.apache.org/jira/browse/KAFKA-1998?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14349128#comment-14349128 ] Guozhang Wang commented on KAFKA-1998: -- This is actually a non-issue, and I was mistaken about the API of partitionsFor() with Jun's recent patch for adding the availablePartitions into Metadata. Closing this ticket. Partitions Missing From MetadataResponse Key: KAFKA-1998 URL: https://issues.apache.org/jira/browse/KAFKA-1998 Project: Kafka Issue Type: Bug Components: core Affects Versions: 0.8.2.0 Reporter: Evan Huus Assignee: Mayuresh Gharat It is known behaviour that when a partition is entirely offline (it has no leader because all of its replicas are down) then that partition will not be included in the metadata returned by other brokers. For example, if topic foo has 3 partitions, but all replicas of partition 3 are offline, then requesting metadata for foo will only return information about partitions 1 and 2. This means that there is no way to reliably determine the number of partitions for a topic via kafka's metadata API; if I receive information on partitions 1 and 2, I don't know if partition 3 is offline or if it is simply that there are only two partitions total. (You can presumably still ask zookeeper directly, but that is a work-around). This ambiguity, in turn, can lead to a consistency problem with the default partitioner, since that effectively implements `hash(key) mod #partitions`. If a partition goes offline and is removed from the metadata response, then the number of partitions the producer knows about will change (on its next metadata refresh) and the mapping from keys to partitions will also change. Instead of distributing messages among (for example) 3 partitions, and failing to produce to the offline partition, it will distribute *all* messages among the two online partitions. This results in messages being sent to the wrong partition. Since kafka already returns partitions with error messages in many cases (e.g. `LeaderNotAvailable`) I think it makes much more sense and fixes the above partition problem if it would simply return offline partitions as well with the appropriate error (whether that is `LeaderNotAvailable` or it would be better to add an additional error is up to you). CC [~guozhang] (This issue was originally described/discussed on the kafka-users mailing list, in the thread involving https://mail-archives.apache.org/mod_mbox/kafka-users/201503.mbox/%3CCAA4pprAZvp2XhdNmy0%2BqVZ1UVdVxmUfz3DDArhGbwP-iiH%2BGyg%40mail.gmail.com%3E) If there are any questions I am happy to clarify, I realize the scenario is somewhat complex. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Commented] (KAFKA-1646) Improve consumer read performance for Windows
[ https://issues.apache.org/jira/browse/KAFKA-1646?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14348401#comment-14348401 ] Honghai Chen commented on KAFKA-1646: - Actually we want to add one more column to file recovery-point-offset-checkpoint, currently it only record offset, like below: 0 2 mvlogs 1 100 mvlogs 0 200 Change to below: 0 2 mvlogs 1 100 8000 mvlogs 0 200 16000 8000 is the start position of the data file for message with offset 100 . And 16000 is start position of the data file for message with offset 200. Take first one as example, when recover the last segment (in function LogSegment.recover(maxMessageSize: Int) , ONLY recover file to min(validBytes, 8000) with offset 100 and rebuild index. Make sense ? [~jkreps] Improve consumer read performance for Windows - Key: KAFKA-1646 URL: https://issues.apache.org/jira/browse/KAFKA-1646 Project: Kafka Issue Type: Improvement Components: log Affects Versions: 0.8.1.1 Environment: Windows Reporter: xueqiang wang Assignee: xueqiang wang Labels: newbie, patch Attachments: Improve consumer read performance for Windows.patch, KAFKA-1646-truncate-off-trailing-zeros-on-broker-restart-if-bro.patch, KAFKA-1646_20141216_163008.patch This patch is for Window platform only. In Windows platform, if there are more than one replicas writing to disk, the segment log files will not be consistent in disk and then consumer reading performance will be dropped down greatly. This fix allocates more disk spaces when rolling a new segment, and then it will improve the consumer reading performance in NTFS file system. This patch doesn't affect file allocation of other filesystems, for it only adds statements like 'if(Os.iswindow)' or adds methods used on Windows. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Commented] (KAFKA-1995) JMS to Kafka: Inbuilt JMSAdaptor/JMSProxy/JMSBridge (Client can speak JMS but hit Kafka)
[ https://issues.apache.org/jira/browse/KAFKA-1995?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14348342#comment-14348342 ] Ewen Cheslack-Postava commented on KAFKA-1995: -- This sounds like a good idea, but I think it doesn't belong in Kafka itself. This sounds a lot like a bunch of other connectors that allow you to pipe data through other systems: https://cwiki.apache.org/confluence/display/KAFKA/Ecosystem I don't know JMS well enough to give suggestions about the implementation, but it seems like the translation of JMS provider - Kafka, JMS producer - Kafka producer and JMS consumer - Kafka consumer should be a pretty minimal, simple layer. You might have better luck getting design feedback on the mailing lists. JMS to Kafka: Inbuilt JMSAdaptor/JMSProxy/JMSBridge (Client can speak JMS but hit Kafka) Key: KAFKA-1995 URL: https://issues.apache.org/jira/browse/KAFKA-1995 Project: Kafka Issue Type: Wish Components: core Affects Versions: 0.8.3 Reporter: Rekha Joshi Kafka is a great alternative to JMS, providing high performance, throughput as scalable, distributed pub sub/commit log service. However there always exist traditional systems running on JMS. Rather than rewriting, it would be great if we just had an inbuilt JMSAdaptor/JMSProxy/JMSBridge by which client can speak JMS but hit Kafka behind-the-scene. Something like Chukwa's o.a.h.chukwa.datacollection.adaptor.jms.JMSAdaptor, which receives msg off JMS queue and transforms to a Chukwa chunk? I have come across folks talking of this need in past as well.Is it considered and/or part of the roadmap? http://grokbase.com/t/kafka/users/131cst8xpv/stomp-binding-for-kafka http://grokbase.com/t/kafka/users/148dm4247q/consuming-messages-from-kafka-and-pushing-on-to-a-jms-queue http://grokbase.com/t/kafka/users/143hjepbn2/request-kafka-zookeeper-jms-details Looking for inputs on correct way to approach this so to retain all good features of Kafka while still not rewriting entire application.Possible? -- This message was sent by Atlassian JIRA (v6.3.4#6332)
Build failed in Jenkins: Kafka-trunk #418
See https://builds.apache.org/job/Kafka-trunk/418/changes Changes: [joe.stein] KAFKA-1845 KafkaConfig should use ConfigDef patch by Andrii Biletskyi reviewed by Gwen Shapira -- [...truncated 1158 lines...] kafka.consumer.TopicFilterTest testBlacklists PASSED kafka.consumer.TopicFilterTest testWildcardTopicCountGetTopicCountMapEscapeJson PASSED kafka.consumer.MetricsTest testMetricsLeak PASSED kafka.consumer.MetricsTest testMetricsReporterAfterDeletingTopic PASSED kafka.zk.ZKEphemeralTest testEphemeralNodeCleanup PASSED kafka.server.ReplicaManagerTest testHighWaterMarkDirectoryMapping PASSED kafka.server.ReplicaManagerTest testHighwaterMarkRelativeDirectoryMapping PASSED kafka.server.ReplicaManagerTest testIllegalRequiredAcks PASSED kafka.server.IsrExpirationTest testIsrExpirationForStuckFollowers PASSED kafka.server.IsrExpirationTest testIsrExpirationForSlowFollowers PASSED kafka.server.SimpleFetchTest testReadFromLog PASSED kafka.server.ServerGenerateBrokerIdTest testAutoGenerateBrokerId PASSED kafka.server.ServerGenerateBrokerIdTest testUserConfigAndGeneratedBrokerId PASSED kafka.server.ServerGenerateBrokerIdTest testMultipleLogDirsMetaProps PASSED kafka.server.ServerGenerateBrokerIdTest testConsistentBrokerIdFromUserConfigAndMetaProps PASSED kafka.server.ServerShutdownTest testCleanShutdown PASSED kafka.server.ServerShutdownTest testCleanShutdownWithDeleteTopicEnabled PASSED kafka.server.ServerShutdownTest testCleanShutdownAfterFailedStartup PASSED kafka.server.ServerShutdownTest testConsecutiveShutdown PASSED kafka.server.KafkaConfigTest testLogRetentionTimeHoursProvided PASSED kafka.server.KafkaConfigTest testLogRetentionTimeMinutesProvided PASSED kafka.server.KafkaConfigTest testLogRetentionTimeMsProvided PASSED kafka.server.KafkaConfigTest testLogRetentionTimeNoConfigProvided PASSED kafka.server.KafkaConfigTest testLogRetentionTimeBothMinutesAndHoursProvided PASSED kafka.server.KafkaConfigTest testLogRetentionTimeBothMinutesAndMsProvided PASSED kafka.server.KafkaConfigTest testAdvertiseDefaults PASSED kafka.server.KafkaConfigTest testAdvertiseConfigured PASSED kafka.server.KafkaConfigTest testUncleanLeaderElectionDefault PASSED kafka.server.KafkaConfigTest testUncleanElectionDisabled PASSED kafka.server.KafkaConfigTest testUncleanElectionEnabled PASSED kafka.server.KafkaConfigTest testUncleanElectionInvalid PASSED kafka.server.KafkaConfigTest testLogRollTimeMsProvided PASSED kafka.server.KafkaConfigTest testLogRollTimeBothMsAndHoursProvided PASSED kafka.server.KafkaConfigTest testLogRollTimeNoConfigProvided PASSED kafka.server.KafkaConfigTest testDefaultCompressionType PASSED kafka.server.KafkaConfigTest testValidCompressionType PASSED kafka.server.KafkaConfigTest testInvalidCompressionType PASSED kafka.server.OffsetCommitTest testUpdateOffsets PASSED kafka.server.OffsetCommitTest testCommitAndFetchOffsets PASSED kafka.server.OffsetCommitTest testLargeMetadataPayload PASSED kafka.server.OffsetCommitTest testNonExistingTopicOffsetCommit PASSED kafka.server.LogOffsetTest testGetOffsetsForUnknownTopic PASSED kafka.server.LogOffsetTest testGetOffsetsBeforeLatestTime PASSED kafka.server.LogOffsetTest testEmptyLogsGetOffsets PASSED kafka.server.LogOffsetTest testGetOffsetsBeforeNow PASSED kafka.server.LogOffsetTest testGetOffsetsBeforeEarliestTime PASSED kafka.server.AdvertiseBrokerTest testBrokerAdvertiseToZK PASSED kafka.server.ServerStartupTest testBrokerCreatesZKChroot PASSED kafka.server.ServerStartupTest testConflictBrokerRegistration PASSED kafka.server.DelayedOperationTest testRequestSatisfaction PASSED kafka.server.DelayedOperationTest testRequestExpiry PASSED kafka.server.DelayedOperationTest testRequestPurge PASSED kafka.server.LeaderElectionTest testLeaderElectionAndEpoch PASSED kafka.server.LeaderElectionTest testLeaderElectionWithStaleControllerEpoch PASSED kafka.server.DynamicConfigChangeTest testConfigChange PASSED kafka.server.DynamicConfigChangeTest testConfigChangeOnNonExistingTopic PASSED kafka.server.HighwatermarkPersistenceTest testHighWatermarkPersistenceSinglePartition PASSED kafka.server.HighwatermarkPersistenceTest testHighWatermarkPersistenceMultiplePartitions PASSED kafka.server.ReplicaFetchTest testReplicaFetcherThread PASSED kafka.server.LogRecoveryTest testHWCheckpointNoFailuresSingleLogSegment PASSED kafka.server.LogRecoveryTest testHWCheckpointWithFailuresSingleLogSegment PASSED kafka.server.LogRecoveryTest testHWCheckpointNoFailuresMultipleLogSegments PASSED kafka.server.LogRecoveryTest testHWCheckpointWithFailuresMultipleLogSegments PASSED kafka.integration.PrimitiveApiTest testFetchRequestCanProperlySerialize PASSED kafka.integration.PrimitiveApiTest testEmptyFetchRequest PASSED kafka.integration.PrimitiveApiTest testDefaultEncoderProducerAndFetch PASSED
[jira] [Created] (KAFKA-2005) Generate html report for system tests
Ashish K Singh created KAFKA-2005: - Summary: Generate html report for system tests Key: KAFKA-2005 URL: https://issues.apache.org/jira/browse/KAFKA-2005 Project: Kafka Issue Type: Improvement Components: system tests Reporter: Ashish K Singh Assignee: Ashish K Singh System test results are kind of huge and painful to read. A html report will be very useful. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Commented] (KAFKA-1995) JMS to Kafka: Inbuilt JMSAdaptor/JMSProxy/JMSBridge (Client can speak JMS but hit Kafka)
[ https://issues.apache.org/jira/browse/KAFKA-1995?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14348951#comment-14348951 ] Joe Stein commented on KAFKA-1995: -- This sounds like a good idea, but I think it doesn't belong in Kafka itself Agreed on both A JMS client would be great https://cwiki.apache.org/confluence/display/KAFKA/Clients for sure. JMS to Kafka: Inbuilt JMSAdaptor/JMSProxy/JMSBridge (Client can speak JMS but hit Kafka) Key: KAFKA-1995 URL: https://issues.apache.org/jira/browse/KAFKA-1995 Project: Kafka Issue Type: Wish Components: core Affects Versions: 0.8.3 Reporter: Rekha Joshi Kafka is a great alternative to JMS, providing high performance, throughput as scalable, distributed pub sub/commit log service. However there always exist traditional systems running on JMS. Rather than rewriting, it would be great if we just had an inbuilt JMSAdaptor/JMSProxy/JMSBridge by which client can speak JMS but hit Kafka behind-the-scene. Something like Chukwa's o.a.h.chukwa.datacollection.adaptor.jms.JMSAdaptor, which receives msg off JMS queue and transforms to a Chukwa chunk? I have come across folks talking of this need in past as well.Is it considered and/or part of the roadmap? http://grokbase.com/t/kafka/users/131cst8xpv/stomp-binding-for-kafka http://grokbase.com/t/kafka/users/148dm4247q/consuming-messages-from-kafka-and-pushing-on-to-a-jms-queue http://grokbase.com/t/kafka/users/143hjepbn2/request-kafka-zookeeper-jms-details Looking for inputs on correct way to approach this so to retain all good features of Kafka while still not rewriting entire application.Possible? -- This message was sent by Atlassian JIRA (v6.3.4#6332)
Re: [DISCUSS] KIP-6 - New reassignment partition logic for re-balancing
I would not think that partitions moving would cause any orphaned messages like that. I would be more concerned about what happens when you change the default on a running cluster from one scheme to another. Would we want to support some kind of automated reassignment of existing partitions (personally - no. I want to trigger that manually because it is a very disk and network intensive process)? -Todd On Wed, Mar 4, 2015 at 7:33 PM, Tong Li liton...@us.ibm.com wrote: Todd, I think plugable design is good with solid default. The only issue I feel is when you use one and switch to another, will we end up with some unread messages hanging around and no one thinks or knows it is their responsibility to take care of them? Thanks. Tong Sent from my iPhone On Mar 5, 2015, at 10:46 AM, Todd Palino tpal...@gmail.com wrote: Apologize for the late comment on this... So fair assignment by count (taking into account the current partition count of each broker) is very good. However, it's worth noting that all partitions are not created equal. We have actually been performing more rebalance work based on the partition size on disk, as given equal retention of all topics, the size on disk is a better indicator of the amount of traffic a partition gets, both in terms of storage and network traffic. Overall, this seems to be a better balance. In addition to this, I think there is very much a need to have Kafka be rack-aware. That is, to be able to assure that for a given cluster, you never assign all replicas for a given partition in the same rack. This would allow us to guard against maintenances or power failures that affect a full rack of systems (or a given switch). I think it would make sense to implement the reassignment logic as a pluggable component. That way it would be easy to select a scheme when performing a reassignment (count, size, rack aware). Configuring a default scheme for a cluster would allow for the brokers to create new topics and partitions in compliance with the requested policy. -Todd On Thu, Jan 22, 2015 at 10:13 PM, Joe Stein joe.st...@stealth.ly wrote: I will go back through the ticket and code and write more up. Should be able to-do that sometime next week. The intention was to not replace existing functionality by issue a WARN on use. The following version it is released we could then deprecate it... I will fix the KIP for that too. On Fri, Jan 23, 2015 at 12:34 AM, Neha Narkhede n...@confluent.io wrote: Hey Joe, 1. Could you add details to the Public Interface section of the KIP? This should include the proposed changes to the partition reassignment tool. Also, maybe the new option can be named --rebalance instead of --re-balance? 2. It makes sense to list --decommission-broker as part of this KIP. Similarly, shouldn't we also have an --add-broker option? The way I see this is that there are several events when a partition reassignment is required. Before this functionality is automated on the broker, the tool will generate an ideal replica placement for each such event. The users should merely have to specify the nature of the event e.g. adding a broker or decommissioning an existing broker or merely rebalancing. 3. If I understand the KIP correctly, the upgrade plan for this feature includes removing the existing --generate option on the partition reassignment tool in 0.8.3 while adding all the new options in the same release. Is that correct? Thanks, Neha On Thu, Jan 22, 2015 at 9:23 PM, Jay Kreps jay.kr...@gmail.com wrote: Ditto on this one. Can you give the algorithm we want to implement? Also I think in terms of scope this is just proposing to change the logic in ReassignPartitionsCommand? I think we've had the discussion various times on the mailing list that what people really want is just for Kafka to do it's best to balance data in an online fashion (for some definition of balance). i.e. if you add a new node partitions would slowly migrate to it, and if a node dies, partitions slowly migrate off it. This could potentially be more work, but I'm not sure how much more. Has anyone thought about how to do it? -Jay On Wed, Jan 21, 2015 at 10:11 PM, Joe Stein joe.st...@stealth.ly wrote: Posted a KIP for --re-balance for partition assignment in reassignment tool. https://cwiki.apache.org/confluence/display/KAFKA/KIP-6+-+New +reassignment+partition+logic+for+re-balancing JIRA https://issues.apache.org/jira/browse/KAFKA-1792 While going through the KIP I thought of one thing from the JIRA that we should change. We should preserve --generate to be existing functionality for the next release it is
[DISCUSS] KIP-11- Authorization design for kafka security
Hi, KIP-11 is open for discussion , I have updated the wiki with the design and open questions. Thanks Parth
RE: [DISCUSS] KIP-6 - New reassignment partition logic for re-balancing
Thanks for the writeup and RB Joe/Dmitry. Is it possible to describe the proposed partition reassignment algorithm in more detail on the KIP? In fact, it would be really easy to understand if we had some concrete examples comparing partition assignment with the old algorithm and the new. Aditya From: Tong Li [liton...@us.ibm.com] Sent: Wednesday, March 04, 2015 7:33 PM To: dev@kafka.apache.org Subject: Re: [DISCUSS] KIP-6 - New reassignment partition logic for re-balancing Todd, I think plugable design is good with solid default. The only issue I feel is when you use one and switch to another, will we end up with some unread messages hanging around and no one thinks or knows it is their responsibility to take care of them? Thanks. Tong Sent from my iPhone On Mar 5, 2015, at 10:46 AM, Todd Palino tpal...@gmail.com wrote: Apologize for the late comment on this... So fair assignment by count (taking into account the current partition count of each broker) is very good. However, it's worth noting that all partitions are not created equal. We have actually been performing more rebalance work based on the partition size on disk, as given equal retention of all topics, the size on disk is a better indicator of the amount of traffic a partition gets, both in terms of storage and network traffic. Overall, this seems to be a better balance. In addition to this, I think there is very much a need to have Kafka be rack-aware. That is, to be able to assure that for a given cluster, you never assign all replicas for a given partition in the same rack. This would allow us to guard against maintenances or power failures that affect a full rack of systems (or a given switch). I think it would make sense to implement the reassignment logic as a pluggable component. That way it would be easy to select a scheme when performing a reassignment (count, size, rack aware). Configuring a default scheme for a cluster would allow for the brokers to create new topics and partitions in compliance with the requested policy. -Todd On Thu, Jan 22, 2015 at 10:13 PM, Joe Stein joe.st...@stealth.ly wrote: I will go back through the ticket and code and write more up. Should be able to-do that sometime next week. The intention was to not replace existing functionality by issue a WARN on use. The following version it is released we could then deprecate it... I will fix the KIP for that too. On Fri, Jan 23, 2015 at 12:34 AM, Neha Narkhede n...@confluent.io wrote: Hey Joe, 1. Could you add details to the Public Interface section of the KIP? This should include the proposed changes to the partition reassignment tool. Also, maybe the new option can be named --rebalance instead of --re-balance? 2. It makes sense to list --decommission-broker as part of this KIP. Similarly, shouldn't we also have an --add-broker option? The way I see this is that there are several events when a partition reassignment is required. Before this functionality is automated on the broker, the tool will generate an ideal replica placement for each such event. The users should merely have to specify the nature of the event e.g. adding a broker or decommissioning an existing broker or merely rebalancing. 3. If I understand the KIP correctly, the upgrade plan for this feature includes removing the existing --generate option on the partition reassignment tool in 0.8.3 while adding all the new options in the same release. Is that correct? Thanks, Neha On Thu, Jan 22, 2015 at 9:23 PM, Jay Kreps jay.kr...@gmail.com wrote: Ditto on this one. Can you give the algorithm we want to implement? Also I think in terms of scope this is just proposing to change the logic in ReassignPartitionsCommand? I think we've had the discussion various times on the mailing list that what people really want is just for Kafka to do it's best to balance data in an online fashion (for some definition of balance). i.e. if you add a new node partitions would slowly migrate to it, and if a node dies, partitions slowly migrate off it. This could potentially be more work, but I'm not sure how much more. Has anyone thought about how to do it? -Jay On Wed, Jan 21, 2015 at 10:11 PM, Joe Stein joe.st...@stealth.ly wrote: Posted a KIP for --re-balance for partition assignment in reassignment tool. https://cwiki.apache.org/confluence/display/KAFKA/KIP-6+-+New +reassignment+partition+logic+for+re-balancing JIRA https://issues.apache.org/jira/browse/KAFKA-1792 While going through the KIP I thought of one thing from the JIRA that we should change. We should preserve --generate to be existing functionality for the next release it is in. If folks want to use --re-balance then great, it just won't break any
Re: [DISCUSS] KIP-11- Authorization design for kafka security
Forgot to add links to wiki and jira. Link to wiki: https://cwiki.apache.org/confluence/display/KAFKA/KIP-11+-+Authorization+Interface Link to Jira: https://issues.apache.org/jira/browse/KAFKA-1688 Thanks Parth From: Parth Brahmbhatt pbrahmbh...@hortonworks.commailto:pbrahmbh...@hortonworks.com Date: Thursday, March 5, 2015 at 10:33 AM To: dev@kafka.apache.orgmailto:dev@kafka.apache.org dev@kafka.apache.orgmailto:dev@kafka.apache.org Subject: [DISCUSS] KIP-11- Authorization design for kafka security Hi, KIP-11 is open for discussion , I have updated the wiki with the design and open questions. Thanks Parth
[jira] [Commented] (KAFKA-1754) KOYA - Kafka on YARN
[ https://issues.apache.org/jira/browse/KAFKA-1754?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14349292#comment-14349292 ] Thomas Weise commented on KAFKA-1754: - @raviprak Slider is the way forward. More info on this: http://hortonworks.com/blog/koya-apache-slider/ KOYA - Kafka on YARN Key: KAFKA-1754 URL: https://issues.apache.org/jira/browse/KAFKA-1754 Project: Kafka Issue Type: New Feature Reporter: Thomas Weise Attachments: DT-KOYA-Proposal- JIRA.pdf YARN (Hadoop 2.x) has enabled clusters to be used for a variety of workloads, emerging as distributed operating system for big data applications. Initiatives are on the way to bring long running services under the YARN umbrella, leveraging it for centralized resource management and operations ([YARN-896] and examples such as HBase, Accumulo or Memcached through Slider). This JIRA is to propose KOYA (Kafka On Yarn), a YARN application master to launch and manage Kafka clusters running on YARN. Brokers will use resources allocated through YARN with support for recovery, monitoring etc. Please see attached for more details. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Issue Comment Deleted] (KAFKA-1754) KOYA - Kafka on YARN
[ https://issues.apache.org/jira/browse/KAFKA-1754?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Thomas Weise updated KAFKA-1754: Comment: was deleted (was: @raviprak Slider is the way forward. More info on this: http://hortonworks.com/blog/koya-apache-slider/ ) KOYA - Kafka on YARN Key: KAFKA-1754 URL: https://issues.apache.org/jira/browse/KAFKA-1754 Project: Kafka Issue Type: New Feature Reporter: Thomas Weise Attachments: DT-KOYA-Proposal- JIRA.pdf YARN (Hadoop 2.x) has enabled clusters to be used for a variety of workloads, emerging as distributed operating system for big data applications. Initiatives are on the way to bring long running services under the YARN umbrella, leveraging it for centralized resource management and operations ([YARN-896] and examples such as HBase, Accumulo or Memcached through Slider). This JIRA is to propose KOYA (Kafka On Yarn), a YARN application master to launch and manage Kafka clusters running on YARN. Brokers will use resources allocated through YARN with support for recovery, monitoring etc. Please see attached for more details. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
Review Request 31765: Patch for KAFKA-2005
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/31765/ --- Review request for kafka. Bugs: KAFKA-2005 https://issues.apache.org/jira/browse/KAFKA-2005 Repository: kafka Description --- KAFKA-2005: Generate html report for system tests Diffs - system_test/system_test_runner.py 5078d4479fab71722751a28c3c8f5c0f61baadec Diff: https://reviews.apache.org/r/31765/diff/ Testing --- Thanks, Ashish Singh
[jira] [Updated] (KAFKA-2005) Generate html report for system tests
[ https://issues.apache.org/jira/browse/KAFKA-2005?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Ashish K Singh updated KAFKA-2005: -- Attachment: KAFKA-2005.patch Generate html report for system tests - Key: KAFKA-2005 URL: https://issues.apache.org/jira/browse/KAFKA-2005 Project: Kafka Issue Type: Improvement Components: system tests Reporter: Ashish K Singh Assignee: Ashish K Singh Attachments: KAFKA-2005.patch System test results are kind of huge and painful to read. A html report will be very useful. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Commented] (KAFKA-2005) Generate html report for system tests
[ https://issues.apache.org/jira/browse/KAFKA-2005?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14348474#comment-14348474 ] Ashish K Singh commented on KAFKA-2005: --- Created reviewboard https://reviews.apache.org/r/31765/ against branch trunk Generate html report for system tests - Key: KAFKA-2005 URL: https://issues.apache.org/jira/browse/KAFKA-2005 Project: Kafka Issue Type: Improvement Components: system tests Reporter: Ashish K Singh Assignee: Ashish K Singh Attachments: KAFKA-2005.patch System test results are kind of huge and painful to read. A html report will be very useful. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Comment Edited] (KAFKA-1646) Improve consumer read performance for Windows
[ https://issues.apache.org/jira/browse/KAFKA-1646?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14348401#comment-14348401 ] Honghai Chen edited comment on KAFKA-1646 at 3/5/15 10:44 AM: -- Hey, [~jkreps] just clarify, the 50MB/s you mentioned before is the checksum calculation on the machine, not copy replica data from other machine, right? If that's true, seemly we need do 3 changes: 1, when call logManager.shutdown. and os is windows , truncate active segment. 2, when start, if the os is windows, add one new segment. 3, remove the change KAFKA-1646-truncate-off-trailing-zeros-on-broker-restart-if-bro.patch made previously since it's unnecessary. Make sense? was (Author: waldenchen): Actually we want to add one more column to file recovery-point-offset-checkpoint, currently it only record offset, like below: 0 2 mvlogs 1 100 mvlogs 0 200 Change to below: 0 2 mvlogs 1 100 8000 mvlogs 0 200 16000 8000 is the start position of the data file for message with offset 100 . And 16000 is start position of the data file for message with offset 200. Take first one as example, when recover the last segment (in function LogSegment.recover(maxMessageSize: Int) , ONLY recover file to min(validBytes, 8000) with offset 100 and rebuild index. Make sense ? [~jkreps] Improve consumer read performance for Windows - Key: KAFKA-1646 URL: https://issues.apache.org/jira/browse/KAFKA-1646 Project: Kafka Issue Type: Improvement Components: log Affects Versions: 0.8.1.1 Environment: Windows Reporter: xueqiang wang Assignee: xueqiang wang Labels: newbie, patch Attachments: Improve consumer read performance for Windows.patch, KAFKA-1646-truncate-off-trailing-zeros-on-broker-restart-if-bro.patch, KAFKA-1646_20141216_163008.patch This patch is for Window platform only. In Windows platform, if there are more than one replicas writing to disk, the segment log files will not be consistent in disk and then consumer reading performance will be dropped down greatly. This fix allocates more disk spaces when rolling a new segment, and then it will improve the consumer reading performance in NTFS file system. This patch doesn't affect file allocation of other filesystems, for it only adds statements like 'if(Os.iswindow)' or adds methods used on Windows. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Updated] (KAFKA-1845) KafkaConfig should use ConfigDef
[ https://issues.apache.org/jira/browse/KAFKA-1845?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Joe Stein updated KAFKA-1845: - Resolution: Fixed Status: Resolved (was: Patch Available) Thanks for the patch Andrii and the review Gwen, committed to trunk. KafkaConfig should use ConfigDef - Key: KAFKA-1845 URL: https://issues.apache.org/jira/browse/KAFKA-1845 Project: Kafka Issue Type: Sub-task Reporter: Gwen Shapira Assignee: Andrii Biletskyi Labels: newbie Fix For: 0.8.3 Attachments: KAFKA-1845.patch, KAFKA-1845_2015-02-08_17:05:22.patch, KAFKA-1845_2015-03-05_01:12:22.patch ConfigDef is already used for the new producer and for TopicConfig. Will be nice to standardize and use one configuration and validation library across the board. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
Re: [DISCUSS] KIP-4 - Command line and centralized administrative operations
Thanks for the updated wiki. A few comments below: 1. Error description in response: I think if some errorCode could indicate several different error cases then we should really change it to multiple codes. In general the errorCode itself would be precise and sufficient for describing the server side errors. 2. Describe topic request: it would be great to go beyond just batching on topic name regex for this request. For example, a very common use case of the topic command is to list all topics whose config A's value is B. With topic name regex then we have to first retrieve __all__ topics's description info and then filter at the client end, which will be a huge burden on ZK. 3. Config K-Vs in create topic: this is related to the previous point; maybe we can add another metadata K-V or just a metadata string along side with config K-V in create topic like we did for offset commit request. This field can be quite useful in storing information like owner of the topic who issue the create command, etc, which is quite important for a multi-tenant setting. Then in the describe topic request we can also batch on regex of the metadata field. 4. Today all the admin operations are async in the sense that command will return once it is written in ZK, and that is why we need extra verification like testUtil.waitForTopicCreated() / verify partition reassignment request, etc. With admin requests we could add a flag to enable / disable synchronous requests; when it is turned on, the response will not return until the request has been completed. And for async requests we can add a token field in the response, and then only need a general admin verification request with the given token to check if the async request has been completed. 5. +1 for extending Metadata request to include controller / coordinator information, and then we can remove the ConsumerMetadata / ClusterMetadata requests. Guozhang On Tue, Mar 3, 2015 at 10:23 AM, Joel Koshy jjkosh...@gmail.com wrote: Thanks for sending that out Joe - I don't think I will be able to make it today, so if notes can be sent out afterward that would be great. On Mon, Mar 02, 2015 at 09:16:13AM -0800, Gwen Shapira wrote: Thanks for sending this out Joe. Looking forward to chatting with everyone :) On Mon, Mar 2, 2015 at 6:46 AM, Joe Stein joe.st...@stealth.ly wrote: Hey, I just sent out a google hangout invite to all pmc, committers and everyone I found working on a KIP. If I missed anyone in the invite please let me know and can update it, np. We should do this every Tuesday @ 2pm Eastern Time. Maybe we can get INFRA help to make a google account so we can manage better? To discuss https://cwiki.apache.org/confluence/display/KAFKA/Kafka+Improvement+Proposals in progress and related JIRA that are interdependent and common work. ~ Joe Stein On Tue, Feb 24, 2015 at 2:59 PM, Jay Kreps jay.kr...@gmail.com wrote: Let's stay on Google hangouts that will also record and make the sessions available on youtube. -Jay On Tue, Feb 24, 2015 at 11:49 AM, Jeff Holoman jholo...@cloudera.com wrote: Jay / Joe We're happy to send out a Webex for this purpose. We could record the sessions if there is interest and publish them out. Thanks Jeff On Tue, Feb 24, 2015 at 11:28 AM, Jay Kreps jay.kr...@gmail.com wrote: Let's try to get the technical hang-ups sorted out, though. I really think there is some benefit to live discussion vs writing. I am hopeful that if we post instructions and give ourselves a few attempts we can get it working. Tuesday at that time would work for me...any objections? -Jay On Tue, Feb 24, 2015 at 8:18 AM, Joe Stein joe.st...@stealth.ly wrote: Weekly would be great maybe like every Tuesday ~ 1pm ET / 10am PT I don't mind google hangout but there is always some issue or whatever so we know the apache irc channel works. We can start there and see how it goes? We can pull transcripts too and associate to tickets if need be makes it helpful for things. ~ Joestein On Tue, Feb 24, 2015 at 11:10 AM, Jay Kreps jay.kr...@gmail.com wrote: We'd talked about doing a Google Hangout to chat about this. What about generalizing that a little further...I actually think it would be good for everyone spending a reasonable chunk of their week on Kafka stuff to maybe sync up once a week. I think we could use time to talk through design stuff, make sure we are on top of code reviews, talk through any tricky issues, etc. We can make it publicly available so that any one can follow along who likes. Any interest in doing this? If so I'll try to set it up starting next week.
Re: Review Request 31706: Patch for KAFKA-1997
On March 6, 2015, 12:14 a.m., Joel Koshy wrote: core/src/main/scala/kafka/consumer/PartitionAssignor.scala, line 74 https://reviews.apache.org/r/31706/diff/4/?file=885021#file885021line74 Why does this need to be a pool? i.e., rebalance is done while holding a lock. It doesn't have to be. But so we can use the methods it provided like getAndMaybePut. It makes code cleaner. On March 6, 2015, 12:14 a.m., Joel Koshy wrote: core/src/main/scala/kafka/consumer/ZookeeperConsumerConnector.scala, line 727 https://reviews.apache.org/r/31706/diff/4/?file=885022#file885022line727 Rather build out the val here, can you move this to a separate line above? Also, we can call it globalPartitionAssignment It is actually different from the globalPartitionAssignment returned by assignor. I changed the name to partitionAssigmentMapForCallback. On March 6, 2015, 12:14 a.m., Joel Koshy wrote: core/src/main/scala/kafka/consumer/ZookeeperConsumerConnector.scala, line 729 https://reviews.apache.org/r/31706/diff/4/?file=885022#file885022line729 partitionOwnerships - topicPartitionAssignment I don't quite get this. Can you elaborate? On March 6, 2015, 12:14 a.m., Joel Koshy wrote: core/src/main/scala/kafka/tools/MirrorMaker.scala, line 241 https://reviews.apache.org/r/31706/diff/4/?file=885024#file885024line241 Why was this change made? It is to avoid that consumer blocks on a low volume pipeline and not able to commit offsets. Just like the timeout for new consumer's poll(). On March 6, 2015, 12:14 a.m., Joel Koshy wrote: core/src/main/scala/kafka/tools/MirrorMaker.scala, line 310 https://reviews.apache.org/r/31706/diff/4/?file=885024#file885024line310 how does the user override? Good catch! - Jiangjie --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/31706/#review75409 --- On March 6, 2015, 4:15 a.m., Jiangjie Qin wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/31706/ --- (Updated March 6, 2015, 4:15 a.m.) Review request for kafka. Bugs: KAFKA-1997 https://issues.apache.org/jira/browse/KAFKA-1997 Repository: kafka Description --- Addressed Guozhang's comments. Changed the exit behavior on send failure because close(0) is not ready yet. Will submit followup patch after KAFKA-1660 is checked in. Expanded imports from _ and * to full class path Incorporated Joel's comments. Diffs - clients/src/main/java/org/apache/kafka/clients/producer/internals/RecordAccumulator.java d5c79e2481d5e9a2524ac2ef6a6879f61cb7cb5f core/src/main/scala/kafka/consumer/PartitionAssignor.scala e6ff7683a0df4a7d221e949767e57c34703d5aad core/src/main/scala/kafka/consumer/ZookeeperConsumerConnector.scala 5487259751ebe19f137948249aa1fd2637d2deb4 core/src/main/scala/kafka/javaapi/consumer/ConsumerRebalanceListener.java 7f45a90ba6676290172b7da54c15ee5dc1a42a2e core/src/main/scala/kafka/tools/MirrorMaker.scala 5374280dc97dc8e01e9b3ba61fd036dc13ae48cb core/src/test/scala/unit/kafka/consumer/PartitionAssignorTest.scala 543070f4fd3e96f3183cae9ee2ccbe843409ee58 core/src/test/scala/unit/kafka/consumer/ZookeeperConsumerConnectorTest.scala a17e8532c44aadf84b8da3a57bcc797a848b5020 Diff: https://reviews.apache.org/r/31706/diff/ Testing --- Thanks, Jiangjie Qin
[jira] [Commented] (KAFKA-1997) Refactor Mirror Maker
[ https://issues.apache.org/jira/browse/KAFKA-1997?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14349930#comment-14349930 ] Jiangjie Qin commented on KAFKA-1997: - Updated reviewboard https://reviews.apache.org/r/31706/diff/ against branch origin/trunk Refactor Mirror Maker - Key: KAFKA-1997 URL: https://issues.apache.org/jira/browse/KAFKA-1997 Project: Kafka Issue Type: Improvement Reporter: Jiangjie Qin Assignee: Jiangjie Qin Attachments: KAFKA-1997.patch, KAFKA-1997_2015-03-03_16:28:46.patch, KAFKA-1997_2015-03-04_15:07:46.patch, KAFKA-1997_2015-03-04_15:42:45.patch, KAFKA-1997_2015-03-05_20:14:58.patch Refactor mirror maker based on KIP-3 -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Updated] (KAFKA-1997) Refactor Mirror Maker
[ https://issues.apache.org/jira/browse/KAFKA-1997?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Jiangjie Qin updated KAFKA-1997: Attachment: KAFKA-1997_2015-03-05_20:14:58.patch Refactor Mirror Maker - Key: KAFKA-1997 URL: https://issues.apache.org/jira/browse/KAFKA-1997 Project: Kafka Issue Type: Improvement Reporter: Jiangjie Qin Assignee: Jiangjie Qin Attachments: KAFKA-1997.patch, KAFKA-1997_2015-03-03_16:28:46.patch, KAFKA-1997_2015-03-04_15:07:46.patch, KAFKA-1997_2015-03-04_15:42:45.patch, KAFKA-1997_2015-03-05_20:14:58.patch Refactor mirror maker based on KIP-3 -- This message was sent by Atlassian JIRA (v6.3.4#6332)
Re: Review Request 31706: Patch for KAFKA-1997
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/31706/ --- (Updated March 6, 2015, 4:15 a.m.) Review request for kafka. Bugs: KAFKA-1997 https://issues.apache.org/jira/browse/KAFKA-1997 Repository: kafka Description (updated) --- Addressed Guozhang's comments. Changed the exit behavior on send failure because close(0) is not ready yet. Will submit followup patch after KAFKA-1660 is checked in. Expanded imports from _ and * to full class path Incorporated Joel's comments. Diffs (updated) - clients/src/main/java/org/apache/kafka/clients/producer/internals/RecordAccumulator.java d5c79e2481d5e9a2524ac2ef6a6879f61cb7cb5f core/src/main/scala/kafka/consumer/PartitionAssignor.scala e6ff7683a0df4a7d221e949767e57c34703d5aad core/src/main/scala/kafka/consumer/ZookeeperConsumerConnector.scala 5487259751ebe19f137948249aa1fd2637d2deb4 core/src/main/scala/kafka/javaapi/consumer/ConsumerRebalanceListener.java 7f45a90ba6676290172b7da54c15ee5dc1a42a2e core/src/main/scala/kafka/tools/MirrorMaker.scala 5374280dc97dc8e01e9b3ba61fd036dc13ae48cb core/src/test/scala/unit/kafka/consumer/PartitionAssignorTest.scala 543070f4fd3e96f3183cae9ee2ccbe843409ee58 core/src/test/scala/unit/kafka/consumer/ZookeeperConsumerConnectorTest.scala a17e8532c44aadf84b8da3a57bcc797a848b5020 Diff: https://reviews.apache.org/r/31706/diff/ Testing --- Thanks, Jiangjie Qin