[jira] [Commented] (KAFKA-687) Rebalance algorithm should consider partitions from all topics
[ https://issues.apache.org/jira/browse/KAFKA-687?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14077480#comment-14077480 ] Joel Koshy commented on KAFKA-687: -- Short update on this: After the initial review comments, I was trying to make the allocation module more generic so we can reuse it in the new consumer. Furthermore, I was trying to get rid of the symmetric mode (which is for wildcards only and with identical subscriptions across all consumers) and make roundrobin more general. The basic approach was to sort the consumer IDs based on a hash of the consumerID with the topic appended to it - effectively scrambling (in a consistent order) the list of consumer streams available for a given topic - and then doing a round-robin assignment across available partitions of the topic. This did not actually work as well as expected. Here is the output of some simulations: {code} [2014-07-25 17:00:35,559] INFO Owned count summary for 6284 partitions across 63 consumer ids (9 consumers with 7 streams): min: 8.00; max: 200.00; avg: 99.746032; stddev: 58.871914; ideal: 99.746033 (unit.kafka.consumer.PartitionAllocatorTest:68) [2014-07-25 17:00:36,791] INFO Owned count summary for 6118 partitions across 42 consumer ids (7 consumers with 6 streams): min: 57.00; max: 254.00; avg: 145.67; stddev: 60.954468; ideal: 145.72 (unit.kafka.consumer.PartitionAllocatorTest:68) [2014-07-25 17:00:38,065] INFO Owned count summary for 10652 partitions across 88 consumer ids (11 consumers with 8 streams): min: 4.00; max: 335.00; avg: 169.079365; stddev: 101.093266; ideal: 121.045456 (unit.kafka.consumer.PartitionAllocatorTest:68) [2014-07-25 17:02:07,198] INFO Owned count summary for 10839 partitions across 200 consumer ids (20 consumers with 10 streams): min: 3.00; max: 330.00; avg: 172.047619; stddev: 99.267223; ideal: 54.195000 (unit.kafka.consumer.PartitionAllocatorTest:68) [2014-07-25 17:24:35,676] INFO Owned count summary for 6439 partitions across 12 consumer ids (2 consumers with 6 streams): min: 445.00; max: 626.00; avg: 536.58; stddev: 58.445714; ideal: 536.583313 (unit.kafka.consumer.PartitionAllocatorTest:68) [2014-07-25 17:24:36,787] INFO Owned count summary for 11777 partitions across 63 consumer ids (7 consumers with 9 streams): min: 5.00; max: 369.00; avg: 186.936508; stddev: 113.972531; ideal: 186.936508 (unit.kafka.consumer.PartitionAllocatorTest:68) [2014-07-25 17:25:20,108] INFO Owned count summary for 10488 partitions across 144 consumer ids (18 consumers with 8 streams): min: 8.00; max: 335.00; avg: 166.476190; stddev: 101.988433; ideal: 72.86 (unit.kafka.consumer.PartitionAllocatorTest:68) [2014-07-25 17:33:52,532] INFO Owned count summary for 5783 partitions across 25 consumer ids (5 consumers with 5 streams): min: 141.00; max: 336.00; avg: 231.32; stddev: 69.337171; ideal: 231.320007 (unit.kafka.consumer.PartitionAllocatorTest:68) [2014-07-25 17:33:53,268] INFO Owned count summary for 6181 partitions across 7 consumer ids (7 consumers with 1 streams): min: 801.00; max: 980.00; avg: 883.00; stddev: 59.654561; ideal: 883.00 (unit.kafka.consumer.PartitionAllocatorTest:68) [2014-07-25 17:33:56,124] INFO Owned count summary for 6475 partitions across 32 consumer ids (4 consumers with 8 streams): min: 105.00; max: 299.00; avg: 202.343750; stddev: 62.999544; ideal: 202.343750 (unit.kafka.consumer.PartitionAllocatorTest:68) [2014-07-25 17:35:10,370] INFO Owned count summary for 7739 partitions across 162 consumer ids (18 consumers with 9 streams): min: 6.00; max: 239.00; avg: 122.841270; stddev: 69.379788; ideal: 47.771606 (unit.kafka.consumer.PartitionAllocatorTest:68) [2014-07-25 17:35:11,834] INFO Owned count summary for 9070 partitions across 14 consumer ids (2 consumers with 7 streams): min: 520.00; max: 774.00; avg: 647.857143; stddev: 84.860843; ideal: 647.857117 (unit.kafka.consumer.PartitionAllocatorTest:68) [2014-07-25 17:36:37,935] INFO Owned count summary for 10933 partitions across 85 consumer ids (17 consumers with 5 streams): min: 5.00; max: 350.00; avg: 173.539683; stddev: 105.619192; ideal: 128.623535 (unit.kafka.consumer.PartitionAllocatorTest:68) [2014-07-25 17:36:40,641] INFO Owned count summary for 8665 partitions across 64 consumer ids (8 consumers with 8 streams): min: 4.00; max: 267.00; avg: 137.539683; stddev: 82.121434; ideal: 135.390625 (unit.kafka.consumer.PartitionAllocatorTest:68) [2014-07-25 17:36:42,612] INFO Owned count summary for 8432 partitions across 48 consumer ids (6 consumers with 8 streams): min: 68.00; max: 328.00; avg: 175.67; stddev: 78.829828; ideal: 175.72 (unit.kafka.consumer.PartitionAllocatorTest:68) [2014-07-25
[jira] [Comment Edited] (KAFKA-1533) transient unit test failure in ProducerFailureHandlingTest
[ https://issues.apache.org/jira/browse/KAFKA-1533?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14077511#comment-14077511 ] David Corley edited comment on KAFKA-1533 at 7/29/14 8:17 AM: -- Hey Jun, I can confirm the test is now passing. However your patch isn't converting the InetAddress to a string representation of the IP. It just needs to be updated to use: {{getInetAddress().getHostAddress()}} was (Author: heavydawson): Hey Jun, I can confirm the test is now passing. However your patch isn't converting the InetAddress to a string representation of the IP. It just needs to be updated to use: getInetAddress().getHostAddress() transient unit test failure in ProducerFailureHandlingTest -- Key: KAFKA-1533 URL: https://issues.apache.org/jira/browse/KAFKA-1533 Project: Kafka Issue Type: Bug Components: core Affects Versions: 0.8.2 Reporter: Jun Rao Assignee: Guozhang Wang Fix For: 0.8.2 Attachments: KAFKA-1533.patch, KAFKA-1533.patch, KAFKA-1533.patch, KAFKA-1533_2014-07-21_15:45:58.patch, kafka.threads, stack.out Occasionally, saw the test hang on tear down. The following is the stack trace. Test worker prio=5 tid=7f9246956000 nid=0x10e078000 in Object.wait() [10e075000] java.lang.Thread.State: WAITING (on object monitor) at java.lang.Object.wait(Native Method) - waiting on 7f4e69578 (a org.apache.zookeeper.ClientCnxn$Packet) at java.lang.Object.wait(Object.java:485) at org.apache.zookeeper.ClientCnxn.submitRequest(ClientCnxn.java:1344) - locked 7f4e69578 (a org.apache.zookeeper.ClientCnxn$Packet) at org.apache.zookeeper.ZooKeeper.delete(ZooKeeper.java:732) at org.I0Itec.zkclient.ZkConnection.delete(ZkConnection.java:91) at org.I0Itec.zkclient.ZkClient$8.call(ZkClient.java:720) at org.I0Itec.zkclient.ZkClient.retryUntilConnected(ZkClient.java:675) at org.I0Itec.zkclient.ZkClient.delete(ZkClient.java:716) at kafka.utils.ZkUtils$.deletePath(ZkUtils.scala:416) at kafka.utils.ZkUtils$.deregisterBrokerInZk(ZkUtils.scala:184) at kafka.server.KafkaHealthcheck.shutdown(KafkaHealthcheck.scala:50) at kafka.server.KafkaServer$$anonfun$shutdown$2.apply$mcV$sp(KafkaServer.scala:243) at kafka.utils.Utils$.swallow(Utils.scala:172) at kafka.utils.Logging$class.swallowWarn(Logging.scala:92) at kafka.utils.Utils$.swallowWarn(Utils.scala:45) at kafka.utils.Logging$class.swallow(Logging.scala:94) at kafka.utils.Utils$.swallow(Utils.scala:45) at kafka.server.KafkaServer.shutdown(KafkaServer.scala:243) at kafka.api.ProducerFailureHandlingTest.tearDown(ProducerFailureHandlingTest.scala:90) -- This message was sent by Atlassian JIRA (v6.2#6252)
[jira] [Commented] (KAFKA-1533) transient unit test failure in ProducerFailureHandlingTest
[ https://issues.apache.org/jira/browse/KAFKA-1533?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14077511#comment-14077511 ] David Corley commented on KAFKA-1533: - Hey Jun, I can confirm the test is now passing. However your patch isn't converting the InetAddress to a string representation of the IP. It just needs to be updated to use: getInetAddress().getHostAddress() transient unit test failure in ProducerFailureHandlingTest -- Key: KAFKA-1533 URL: https://issues.apache.org/jira/browse/KAFKA-1533 Project: Kafka Issue Type: Bug Components: core Affects Versions: 0.8.2 Reporter: Jun Rao Assignee: Guozhang Wang Fix For: 0.8.2 Attachments: KAFKA-1533.patch, KAFKA-1533.patch, KAFKA-1533.patch, KAFKA-1533_2014-07-21_15:45:58.patch, kafka.threads, stack.out Occasionally, saw the test hang on tear down. The following is the stack trace. Test worker prio=5 tid=7f9246956000 nid=0x10e078000 in Object.wait() [10e075000] java.lang.Thread.State: WAITING (on object monitor) at java.lang.Object.wait(Native Method) - waiting on 7f4e69578 (a org.apache.zookeeper.ClientCnxn$Packet) at java.lang.Object.wait(Object.java:485) at org.apache.zookeeper.ClientCnxn.submitRequest(ClientCnxn.java:1344) - locked 7f4e69578 (a org.apache.zookeeper.ClientCnxn$Packet) at org.apache.zookeeper.ZooKeeper.delete(ZooKeeper.java:732) at org.I0Itec.zkclient.ZkConnection.delete(ZkConnection.java:91) at org.I0Itec.zkclient.ZkClient$8.call(ZkClient.java:720) at org.I0Itec.zkclient.ZkClient.retryUntilConnected(ZkClient.java:675) at org.I0Itec.zkclient.ZkClient.delete(ZkClient.java:716) at kafka.utils.ZkUtils$.deletePath(ZkUtils.scala:416) at kafka.utils.ZkUtils$.deregisterBrokerInZk(ZkUtils.scala:184) at kafka.server.KafkaHealthcheck.shutdown(KafkaHealthcheck.scala:50) at kafka.server.KafkaServer$$anonfun$shutdown$2.apply$mcV$sp(KafkaServer.scala:243) at kafka.utils.Utils$.swallow(Utils.scala:172) at kafka.utils.Logging$class.swallowWarn(Logging.scala:92) at kafka.utils.Utils$.swallowWarn(Utils.scala:45) at kafka.utils.Logging$class.swallow(Logging.scala:94) at kafka.utils.Utils$.swallow(Utils.scala:45) at kafka.server.KafkaServer.shutdown(KafkaServer.scala:243) at kafka.api.ProducerFailureHandlingTest.tearDown(ProducerFailureHandlingTest.scala:90) -- This message was sent by Atlassian JIRA (v6.2#6252)
[jira] [Commented] (KAFKA-1502) source jar is empty
[ https://issues.apache.org/jira/browse/KAFKA-1502?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14077631#comment-14077631 ] Ivan Balashov commented on KAFKA-1502: -- http://repo1.maven.org/maven2/org/apache/kafka/kafka_2.10/0.8.1.1/kafka_2.10-0.8.1.1-sources.jar Still empty, any ideas how to fix? source jar is empty --- Key: KAFKA-1502 URL: https://issues.apache.org/jira/browse/KAFKA-1502 Project: Kafka Issue Type: Bug Components: build Affects Versions: 0.8.2 Reporter: Jun Rao Assignee: Joel Koshy Labels: newbie When doing a local publish, kafka_2.8.0-0.8.1.1-sources.jar only contains the following files. META-INF/ META-INF/MANIFEST.MF LICENSE NOTICE -- This message was sent by Atlassian JIRA (v6.2#6252)
[jira] [Commented] (KAFKA-1542) normal IOException in the new producer is logged as ERROR
[ https://issues.apache.org/jira/browse/KAFKA-1542?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14077741#comment-14077741 ] David Corley commented on KAFKA-1542: - Strictly speaking, toString can return either the hostname and the ipaddress [http://docs.oracle.com/javase/1.5.0/docs/api/java/net/InetAddress.html#toString()], whereas getHostAddress will always be just the IP address. That said, I defer to you guys on this. It was you who request the host info, so happy to run with your suggestion. normal IOException in the new producer is logged as ERROR - Key: KAFKA-1542 URL: https://issues.apache.org/jira/browse/KAFKA-1542 Project: Kafka Issue Type: Bug Affects Versions: 0.8.2 Reporter: Jun Rao Assignee: David Corley Labels: newbie Fix For: 0.8.2 Attachments: KAFKA-1542.patch Saw the following error in the log. It seems this can happen if the broker is down. So, this probably should be logged as WARN, instead ERROR. 2014/07/16 00:12:51.799 [Selector] Error in I/O: java.io.IOException: Connection timed out at sun.nio.ch.FileDispatcherImpl.read0(Native Method) at sun.nio.ch.SocketDispatcher.read(SocketDispatcher.java:39) at sun.nio.ch.IOUtil.readIntoNativeBuffer(IOUtil.java:223) at sun.nio.ch.IOUtil.read(IOUtil.java:197) at sun.nio.ch.SocketChannelImpl.read(SocketChannelImpl.java:379) at org.apache.kafka.common.network.NetworkReceive.readFrom(NetworkReceive.java:60) at org.apache.kafka.common.network.Selector.poll(Selector.java:241) at org.apache.kafka.clients.NetworkClient.poll(NetworkClient.java:171) at org.apache.kafka.clients.producer.internals.Sender.run(Sender.java:174) at org.apache.kafka.clients.producer.internals.Sender.run(Sender.java:114) at java.lang.Thread.run(Thread.java:744) -- This message was sent by Atlassian JIRA (v6.2#6252)
[jira] [Commented] (KAFKA-1542) normal IOException in the new producer is logged as ERROR
[ https://issues.apache.org/jira/browse/KAFKA-1542?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14077780#comment-14077780 ] Jun Rao commented on KAFKA-1542: Yes, I realized that the InetAddress can be null. So, instead of doing another null check, it's simpler to just print out itself. normal IOException in the new producer is logged as ERROR - Key: KAFKA-1542 URL: https://issues.apache.org/jira/browse/KAFKA-1542 Project: Kafka Issue Type: Bug Affects Versions: 0.8.2 Reporter: Jun Rao Assignee: David Corley Labels: newbie Fix For: 0.8.2 Attachments: KAFKA-1542.patch Saw the following error in the log. It seems this can happen if the broker is down. So, this probably should be logged as WARN, instead ERROR. 2014/07/16 00:12:51.799 [Selector] Error in I/O: java.io.IOException: Connection timed out at sun.nio.ch.FileDispatcherImpl.read0(Native Method) at sun.nio.ch.SocketDispatcher.read(SocketDispatcher.java:39) at sun.nio.ch.IOUtil.readIntoNativeBuffer(IOUtil.java:223) at sun.nio.ch.IOUtil.read(IOUtil.java:197) at sun.nio.ch.SocketChannelImpl.read(SocketChannelImpl.java:379) at org.apache.kafka.common.network.NetworkReceive.readFrom(NetworkReceive.java:60) at org.apache.kafka.common.network.Selector.poll(Selector.java:241) at org.apache.kafka.clients.NetworkClient.poll(NetworkClient.java:171) at org.apache.kafka.clients.producer.internals.Sender.run(Sender.java:174) at org.apache.kafka.clients.producer.internals.Sender.run(Sender.java:114) at java.lang.Thread.run(Thread.java:744) -- This message was sent by Atlassian JIRA (v6.2#6252)
Re: Review Request 23895: Patch for KAFKA-1419
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/23895/ --- (Updated July 29, 2014, 3:13 p.m.) Review request for kafka. Bugs: KAFKA-1419 https://issues.apache.org/jira/browse/KAFKA-1419 Repository: kafka Description (updated) --- KAFKA-1419 - cross build for scala 2.11 - dropped scala 2.8 support - minor bug fixes KAFKA-1419 - cross build for scala 2.11 - changed 2.11 specific dependency version - updated scala version to 2.11.2 - added getBuffer to ByteBufferMessageSet classes Diffs (updated) - build.gradle a72905df824ba68bed5d5170d18873c23e1782c9 core/src/main/scala/kafka/javaapi/message/ByteBufferMessageSet.scala fecee8d5f7b32f483bb1bfc6a5080d589906f9c4 core/src/main/scala/kafka/message/ByteBufferMessageSet.scala 73401c5ff34d08abce22267aa9c4d86632c6fb74 gradle.properties 4827769a3f8e34f0fe7e783eb58e44d4db04859b gradle/buildscript.gradle 225e0a82708bc5f390e5e2c1d4d9a0d06f491b95 gradle/wrapper/gradle-wrapper.properties 610282a699afc89a82203ef0e4e71ecc53761039 scala.gradle ebd21b870c0746aade63248344ab65d9b5baf820 Diff: https://reviews.apache.org/r/23895/diff/ Testing --- Thanks, Ivan Lyutov
[jira] [Commented] (KAFKA-1419) cross build for scala 2.11
[ https://issues.apache.org/jira/browse/KAFKA-1419?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14077817#comment-14077817 ] Ivan Lyutov commented on KAFKA-1419: Updated reviewboard https://reviews.apache.org/r/23895/diff/ against branch apache/trunk cross build for scala 2.11 -- Key: KAFKA-1419 URL: https://issues.apache.org/jira/browse/KAFKA-1419 Project: Kafka Issue Type: Improvement Components: clients Affects Versions: 0.8.1 Reporter: Scott Clasen Assignee: Ivan Lyutov Priority: Blocker Fix For: 0.8.2 Attachments: KAFKA-1419.patch, KAFKA-1419.patch, KAFKA-1419_2014-07-28_15:05:16.patch, KAFKA-1419_2014-07-29_15:13:43.patch Please publish builds for scala 2.11, hopefully just needs a small tweak to the gradle conf? -- This message was sent by Atlassian JIRA (v6.2#6252)
[jira] [Updated] (KAFKA-1419) cross build for scala 2.11
[ https://issues.apache.org/jira/browse/KAFKA-1419?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Ivan Lyutov updated KAFKA-1419: --- Attachment: KAFKA-1419_2014-07-29_15:13:43.patch cross build for scala 2.11 -- Key: KAFKA-1419 URL: https://issues.apache.org/jira/browse/KAFKA-1419 Project: Kafka Issue Type: Improvement Components: clients Affects Versions: 0.8.1 Reporter: Scott Clasen Assignee: Ivan Lyutov Priority: Blocker Fix For: 0.8.2 Attachments: KAFKA-1419.patch, KAFKA-1419.patch, KAFKA-1419_2014-07-28_15:05:16.patch, KAFKA-1419_2014-07-29_15:13:43.patch Please publish builds for scala 2.11, hopefully just needs a small tweak to the gradle conf? -- This message was sent by Atlassian JIRA (v6.2#6252)
[jira] [Commented] (KAFKA-865) Mavenize and separate the client.
[ https://issues.apache.org/jira/browse/KAFKA-865?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14078121#comment-14078121 ] Jay Kreps commented on KAFKA-865: - The new producer has been completed and the consumer is underway. Mavenize and separate the client. - Key: KAFKA-865 URL: https://issues.apache.org/jira/browse/KAFKA-865 Project: Kafka Issue Type: Improvement Components: clients Affects Versions: 0.8.0 Reporter: Ashwanth Fernando It seems that the java client for Kafka is also bundled with the server JAR file and this is generated using sbt package. This is difficult for java folks to work with because: 1) Many java shops use maven and they want to specify the GAV of kafka in their pom and bang, the client jar and all its dependencies should be added to the application's classpath. I can't do that right now, because I need to run ./sbt eclipse, get the .JAR, add that to my classpath, add a whole lot of dependencies (log4j, slf4j, zkClient and so on) manually, which is a pain. There are 90 million maven central uploads/downloads in 2012 alone. Almost all the java shops out there have maven (either central or in house sonatype). 2) Separation of concerns - keeping the server (core) and the client's classes together in same jar file, increases the size of the bundle for a client and also everytime the server's code changes and a release is performed, the client also needs to update their .JAR file. which is not very great. We don't want a ton of clients to update their .JAR file, just because a faster replication strategy for the kafka server cluster changed in a new release. Action items are to separate the client and server portions of Kafka, add it in a pom along with the compile time dependencies and upload it to Maven Central or if you have a LinkedIn externally exposed Nexus, over there. This will increase adoption of the Kafka framework. -- This message was sent by Atlassian JIRA (v6.2#6252)
[jira] [Created] (KAFKA-1562) kafka-topics.sh alter add partitions resets cleanup.policy
Kenny created KAFKA-1562: Summary: kafka-topics.sh alter add partitions resets cleanup.policy Key: KAFKA-1562 URL: https://issues.apache.org/jira/browse/KAFKA-1562 Project: Kafka Issue Type: Bug Affects Versions: 0.8.1.1 Reporter: Kenny When partitions are added to an already existing topic the cleanup.policy=compact is not retained. {code} ./kafka-topics.sh --zookeeper localhost --create --partitions 1 --replication-factor 1 --topic KTEST --config cleanup.policy=compact ./kafka-topics.sh --zookeeper localhost --describe --topic KTEST Topic:KTEST PartitionCount:1ReplicationFactor:1 Configs:cleanup.policy=compact Topic: KTESTPartition: 0Leader: 0 Replicas: 0 Isr: 0 ./kafka-topics.sh --zookeeper localhost --alter --partitions 3 --topic KTEST --config cleanup.policy=compact ./kafka-topics.sh --zookeeper localhost --describe --topic KTEST Topic:KTEST PartitionCount:3ReplicationFactor:1 Configs: Topic: KTESTPartition: 0Leader: 0 Replicas: 0 Isr: 0 Topic: KTESTPartition: 1Leader: 0 Replicas: 0 Isr: 0 Topic: KTESTPartition: 2Leader: 0 Replicas: 0 Isr: 0 {code} -- This message was sent by Atlassian JIRA (v6.2#6252)
[jira] [Comment Edited] (KAFKA-1477) add authentication layer and initial JKS x509 implementation for brokers, producers and consumer for network communication
[ https://issues.apache.org/jira/browse/KAFKA-1477?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14077832#comment-14077832 ] Joe Stein edited comment on KAFKA-1477 at 7/29/14 7:50 PM: --- Hi Jun, maybe what we (myself, Ivan and the other developers working @ BDOSS) should do then is keep this patch up to date (from a rebase / fix perspective) so folks that are using it already (as there are some folks doing that) and folks that can't use Kafka with out it (there are folks in that camp too) and continue to keep it updated so they still get the benefits coming in 0.8.2 (and moving onwards until/if it gets upstream). It requires some more work on our part and on theirs but that is the trade off we would have to accept. Then we can add to the design doc as you suggest and take changes that come up from there and work them back into the patch (or create a new one) as appropriate and release it as the team can agree for the community needs. Another option to the dangling patch approach (which I have seen be an issue in projects) is a security branch. This approach I have seen be problematic from a community perspective especially with voting and releasing. I am not sure if it was the project team members that caused this or the approach they took or something else, unsure. I would be cautious with going the branch route and I don't know dunno if it would be better but maybe? I also don't know if there were enough other pmc members that would vote for a branch release (regardless of what it was) and then also if they wold vote these changes in a branch release or what folks think of this in general. Having something available from an Apache release perspective has certain usefulness within organizations that you can't get any other way. From my perspective I want to-do what is going to be best for the community and the project. Personally I am happy to spend my time and commit BDOSS resources to apply the patch when we need to for our use or our clients need for it... I can't speak for others though, Per the port - there may be use case(s) that you need to have both the secure and non secure port on at the same time. Maybe what we do is make it configurable so you can turn off the none secure port along with enabling a secure port or even enable both. I know having only the secure and authenticated port on is a use case. was (Author: joestein): Hi Jun, maybe what we (myself, Ivan and the other developers working @ BDOSS) should do then is keep this patch up to date (from a rebase / fix perspective) so folks that are using it already (as there are some folks doing that) and folks that can't use Kafka with out it (there are folks in that camp too) and continue to keep it updated so they still get the benefits coming in 0.8.2 (and moving onwards until/if it gets upstream). It requires some more work on our part and on theirs but that is the trade off we would have to accept. Then we can add to the design doc as you suggest and take changes that come up from there and work them back into the patch (or create a new one) as appropriate and release it as the team can agree for the community needs. Another option to the dangling patch approach (which I have seen be an issue in projects) is a security branch. This approach I have seen be problematic from a community perspective especially with voting and releasing. I am not sure if it was the project team members that caused this or the approach they took or something else, unsure. I would be cautious with going the branch route and I don't know dunno if it would be better but maybe? I also don't know if there were enough other pmc members that would vote for a branch release (regardless of what it was) and then also if they wold vote these changes in a branch release or what folks think of this in general. Having something available from an Apache perspective release perspective has certain usefulness/requirements within organizations that you can't get any other way. From my perspective I want to-do what is going to be best for the community and the project. Personally I am happy to spend my time and commit BDOSS resources to apply the patch when we need to for our use or our clients need for it... I can't speak for others though, Per the port - there may be use case(s) that you need to have both the secure and non secure ports on so maybe what we do is make it configurable so you can turn off the none secure port along with enabling a secure port. I know having only a secure and authenticated port on is a use case. add authentication layer and initial JKS x509 implementation for brokers, producers and consumer for network communication -- Key:
[jira] [Commented] (KAFKA-777) Add system tests for important tools
[ https://issues.apache.org/jira/browse/KAFKA-777?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14078433#comment-14078433 ] Nodir Yuldashev commented on KAFKA-777: --- Created reviewboard against branch origin/trunk Add system tests for important tools Key: KAFKA-777 URL: https://issues.apache.org/jira/browse/KAFKA-777 Project: Kafka Issue Type: Bug Affects Versions: 0.8.0 Reporter: Sriram Subramanian Assignee: John Fung Labels: kafka-0.8, p2, replication-testing Fix For: 0.9.0 Attachments: KAFKA-777.patch Few tools were broken after the zk format change. It would be great to catch these issues during system tests. Some of the tools are 1. ShudownBroker 2. PreferredReplicaAssignment 3. ConsumerOffsetChecker There might be a few more for which we need tests. Need to add them once identified. -- This message was sent by Atlassian JIRA (v6.2#6252)
[jira] [Commented] (KAFKA-1477) add authentication layer and initial JKS x509 implementation for brokers, producers and consumer for network communication
[ https://issues.apache.org/jira/browse/KAFKA-1477?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14078438#comment-14078438 ] Jay Kreps commented on KAFKA-1477: -- Yeah Hey Joe, that is a great offer. I don't personally have a big preference between patch and branch since either way I suppose you end up managing it via git. I don't think there is any concern about upgrading the old producer and consumer. If folks are willing to do that, and we have sufficient test coverage for the security related stuff, then that is great. To second what Jun is saying I think for big user-facing stuff like this it is good if we can avoid incremental development. Even though that is good for us developers, I think it can be frustrating for infrastructure users if things keep churning under them. So I think what we need to do is what Jun described. We need to get a document together that fully describes the cases we need to support, and how people will use these features, as well as the relevant details of implementation and how we will handle ongoing testing in this area. This will let us have something to circulate to get broad consensus among users, who have very different environments, so that what we build will work for all of them (or at least the subset that makes sense). When we have this figured out I think it may well prove possible to take pieces of functionality a bit at a time as we will know where we are going and not have to worry about doing and then redoing things as we evolve our approach. I think the stakeholders here are at least the people who have expressed interest so far which to my knowledge is BDOSS, Salesforce, LinkedIn, Cloudera, and Hortonworks. But I expect there are a substantial number of others who would have input. I'd like to work with you to flesh out these requirements more. I am pretty overbooked this week, but should be more free to start next week if that would work. add authentication layer and initial JKS x509 implementation for brokers, producers and consumer for network communication -- Key: KAFKA-1477 URL: https://issues.apache.org/jira/browse/KAFKA-1477 Project: Kafka Issue Type: New Feature Reporter: Joe Stein Assignee: Ivan Lyutov Fix For: 0.8.2 Attachments: KAFKA-1477-binary.patch, KAFKA-1477.patch, KAFKA-1477_2014-06-02_16:59:40.patch, KAFKA-1477_2014-06-02_17:24:26.patch, KAFKA-1477_2014-06-03_13:46:17.patch -- This message was sent by Atlassian JIRA (v6.2#6252)
[jira] [Updated] (KAFKA-777) Add system tests for important tools
[ https://issues.apache.org/jira/browse/KAFKA-777?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Nodir Yuldashev updated KAFKA-777: -- Attachment: KAFKA-777.patch Add system tests for important tools Key: KAFKA-777 URL: https://issues.apache.org/jira/browse/KAFKA-777 Project: Kafka Issue Type: Bug Affects Versions: 0.8.0 Reporter: Sriram Subramanian Assignee: John Fung Labels: kafka-0.8, p2, replication-testing Fix For: 0.9.0 Attachments: KAFKA-777.patch Few tools were broken after the zk format change. It would be great to catch these issues during system tests. Some of the tools are 1. ShudownBroker 2. PreferredReplicaAssignment 3. ConsumerOffsetChecker There might be a few more for which we need tests. Need to add them once identified. -- This message was sent by Atlassian JIRA (v6.2#6252)
[jira] [Commented] (KAFKA-777) Add system tests for important tools
[ https://issues.apache.org/jira/browse/KAFKA-777?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14078445#comment-14078445 ] Nodir Yuldashev commented on KAFKA-777: --- Created reviewboard against branch origin/trunk Add system tests for important tools Key: KAFKA-777 URL: https://issues.apache.org/jira/browse/KAFKA-777 Project: Kafka Issue Type: Bug Affects Versions: 0.8.0 Reporter: Sriram Subramanian Assignee: John Fung Labels: kafka-0.8, p2, replication-testing Fix For: 0.9.0 Attachments: KAFKA-777.patch, KAFKA-777.patch Few tools were broken after the zk format change. It would be great to catch these issues during system tests. Some of the tools are 1. ShudownBroker 2. PreferredReplicaAssignment 3. ConsumerOffsetChecker There might be a few more for which we need tests. Need to add them once identified. -- This message was sent by Atlassian JIRA (v6.2#6252)
[jira] [Updated] (KAFKA-777) Add system tests for important tools
[ https://issues.apache.org/jira/browse/KAFKA-777?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Nodir Yuldashev updated KAFKA-777: -- Attachment: KAFKA-777.patch Add system tests for important tools Key: KAFKA-777 URL: https://issues.apache.org/jira/browse/KAFKA-777 Project: Kafka Issue Type: Bug Affects Versions: 0.8.0 Reporter: Sriram Subramanian Assignee: John Fung Labels: kafka-0.8, p2, replication-testing Fix For: 0.9.0 Attachments: KAFKA-777.patch, KAFKA-777.patch Few tools were broken after the zk format change. It would be great to catch these issues during system tests. Some of the tools are 1. ShudownBroker 2. PreferredReplicaAssignment 3. ConsumerOffsetChecker There might be a few more for which we need tests. Need to add them once identified. -- This message was sent by Atlassian JIRA (v6.2#6252)
[jira] [Commented] (KAFKA-1477) add authentication layer and initial JKS x509 implementation for brokers, producers and consumer for network communication
[ https://issues.apache.org/jira/browse/KAFKA-1477?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14078452#comment-14078452 ] Rajasekar Elango commented on KAFKA-1477: - Very Good Idea [~joestein] . We (@ salesforce) like secure features to be rebased with latest release so that we can get benefits of using latest version. if required we can help with merging/testing etc. In parallel, we should also plan to merge this to trunk (after design spec review) to avoid cost of merging testing for each rebase. For the port, our use case is to run it in secure mode or in non-secure mode. We thought about supporting both secure and non-secure at different ports, but it looked more complicated to implement, so we went with simple flag to turn on secure mode. Thanks, Raja. add authentication layer and initial JKS x509 implementation for brokers, producers and consumer for network communication -- Key: KAFKA-1477 URL: https://issues.apache.org/jira/browse/KAFKA-1477 Project: Kafka Issue Type: New Feature Reporter: Joe Stein Assignee: Ivan Lyutov Fix For: 0.8.2 Attachments: KAFKA-1477-binary.patch, KAFKA-1477.patch, KAFKA-1477_2014-06-02_16:59:40.patch, KAFKA-1477_2014-06-02_17:24:26.patch, KAFKA-1477_2014-06-03_13:46:17.patch -- This message was sent by Atlassian JIRA (v6.2#6252)
Re: Review Request 24006: Patch for KAFKA-1420
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/24006/#review49049 --- core/src/test/scala/unit/kafka/admin/AdminTest.scala https://reviews.apache.org/r/24006/#comment85879 Could you group kafka imports together before java/scala/other-libs imports? core/src/test/scala/unit/kafka/admin/AdminTest.scala https://reviews.apache.org/r/24006/#comment85878 Could we use val numPartitions = 12 val replicationFactor = 3 and then create expectedReplicaAssignment and leaderForPartitionMap based on these two variables, and re-use them here? core/src/test/scala/unit/kafka/admin/AdminTest.scala https://reviews.apache.org/r/24006/#comment85880 expectedReplicaAssignment seems not used any more. core/src/test/scala/unit/kafka/admin/AdminTest.scala https://reviews.apache.org/r/24006/#comment85888 Could you add a comment here for bouncing server 1? core/src/test/scala/unit/kafka/admin/AdminTest.scala https://reviews.apache.org/r/24006/#comment85890 Is there a specific reason we want to use 10 seconds instead of default 5 seconds? core/src/test/scala/unit/kafka/admin/AdminTest.scala https://reviews.apache.org/r/24006/#comment85889 Is this println intended? core/src/test/scala/unit/kafka/admin/DeleteTopicTest.scala https://reviews.apache.org/r/24006/#comment85891 Do we still need expectedReplicaAssignment? core/src/test/scala/unit/kafka/utils/TestUtils.scala https://reviews.apache.org/r/24006/#comment85892 Could we just set the default value of configs parameter to null, instead of creating a separate function? - Guozhang Wang On July 28, 2014, 8:52 p.m., Jonathan Natkins wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/24006/ --- (Updated July 28, 2014, 8:52 p.m.) Review request for kafka. Bugs: KAFKA-1420 https://issues.apache.org/jira/browse/KAFKA-1420 Repository: kafka Description --- KAFKA-1420 Replace AdminUtils.createOrUpdateTopicPartitionAssignmentPathInZK with TestUtils.createTopic in all unit tests Diffs - core/src/test/scala/unit/kafka/admin/AdminTest.scala e28979827110dfbbb92fe5b152e7f1cc973de400 core/src/test/scala/unit/kafka/admin/DeleteTopicTest.scala 29cc01bcef9cacd8dec1f5d662644fc6fe4994bc core/src/test/scala/unit/kafka/integration/UncleanLeaderElectionTest.scala f44568cb25edf25db857415119018fd4c9922f61 core/src/test/scala/unit/kafka/utils/TestUtils.scala c4e13c5240c8303853d08cc3b40088f8c7dae460 Diff: https://reviews.apache.org/r/24006/diff/ Testing --- Automated Thanks, Jonathan Natkins
[jira] [Commented] (KAFKA-1374) LogCleaner (compaction) does not support compressed topics
[ https://issues.apache.org/jira/browse/KAFKA-1374?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14078858#comment-14078858 ] Manikumar Reddy commented on KAFKA-1374: I am trying to look in to the issue and WIP Patch. we are using the following code to traverse segment offsets . {code} var currOffset = segment.baseOffset while (currOffset segment.index.lastOffset) { currOffset = entry.nextOffset } {code} As per my observation, segment.index.lastOffset is not giving the last offset of a given segment. I have a segment with startingOffset=0 and lastOffset=7140. I am getting segment.index.lastOffset=7118. This is creating some issue in the code. Any idea on why segment.index.lastOffset is not returning proper lastOffset.? LogCleaner (compaction) does not support compressed topics -- Key: KAFKA-1374 URL: https://issues.apache.org/jira/browse/KAFKA-1374 Project: Kafka Issue Type: Bug Reporter: Joel Koshy Labels: newbie++ Fix For: 0.8.2 This is a known issue, but opening a ticket to track. If you try to compact a topic that has compressed messages you will run into various exceptions - typically because during iteration we advance the position based on the decompressed size of the message. I have a bunch of stack traces, but it should be straightforward to reproduce. -- This message was sent by Atlassian JIRA (v6.2#6252)
[jira] [Comment Edited] (KAFKA-1374) LogCleaner (compaction) does not support compressed topics
[ https://issues.apache.org/jira/browse/KAFKA-1374?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14078858#comment-14078858 ] Manikumar Reddy edited comment on KAFKA-1374 at 7/30/14 4:29 AM: - I am trying to look in to the issue and WIP Patch. In WIP patch, the following code is used to traverse segment offsets . {code} var currOffset = segment.baseOffset while (currOffset segment.index.lastOffset) { currOffset = entry.nextOffset } {code} As per my observation, segment.index.lastOffset is not giving the last offset of a given segment. I have a segment with startingOffset=0 and lastOffset=7140. I am getting segment.index.lastOffset=7118. This is creating some issue in the code. Any idea on why segment.index.lastOffset is not returning proper lastOffset.? was (Author: omkreddy): I am trying to look in to the issue and WIP Patch. we are using the following code to traverse segment offsets . {code} var currOffset = segment.baseOffset while (currOffset segment.index.lastOffset) { currOffset = entry.nextOffset } {code} As per my observation, segment.index.lastOffset is not giving the last offset of a given segment. I have a segment with startingOffset=0 and lastOffset=7140. I am getting segment.index.lastOffset=7118. This is creating some issue in the code. Any idea on why segment.index.lastOffset is not returning proper lastOffset.? LogCleaner (compaction) does not support compressed topics -- Key: KAFKA-1374 URL: https://issues.apache.org/jira/browse/KAFKA-1374 Project: Kafka Issue Type: Bug Reporter: Joel Koshy Labels: newbie++ Fix For: 0.8.2 This is a known issue, but opening a ticket to track. If you try to compact a topic that has compressed messages you will run into various exceptions - typically because during iteration we advance the position based on the decompressed size of the message. I have a bunch of stack traces, but it should be straightforward to reproduce. -- This message was sent by Atlassian JIRA (v6.2#6252)
[jira] [Commented] (KAFKA-1510) Force offset commits when migrating consumer offsets from zookeeper to kafka
[ https://issues.apache.org/jira/browse/KAFKA-1510?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14078904#comment-14078904 ] nicu marasoiu commented on KAFKA-1510: -- [~jjkoshy] Can you please take a look at my comments+code, it will probably take one more iteration at least to make it. Force offset commits when migrating consumer offsets from zookeeper to kafka Key: KAFKA-1510 URL: https://issues.apache.org/jira/browse/KAFKA-1510 Project: Kafka Issue Type: Bug Affects Versions: 0.8.2 Reporter: Joel Koshy Assignee: Joel Koshy Labels: newbie Fix For: 0.8.2 Attachments: forceCommitOnShutdownWhenDualCommit.patch When migrating consumer offsets from ZooKeeper to kafka, we have to turn on dual-commit (i.e., the consumers will commit offsets to both zookeeper and kafka) in addition to setting offsets.storage to kafka. However, when we commit offsets we only commit offsets if they have changed (since the last commit). For low-volume topics or for topics that receive data in bursts offsets may not move for a long period of time. Therefore we may want to force the commit (even if offsets have not changed) when migrating (i.e., when dual-commit is enabled) - we can add a minimum interval threshold (say force commit after every 10 auto-commits) as well as on rebalance and shutdown. Also, I think it is safe to switch the default for offsets.storage from zookeeper to kafka and set the default to dual-commit (for people who have not migrated yet). We have deployed this to the largest consumers at linkedin and have not seen any issues so far (except for the migration caveat that this jira will resolve). -- This message was sent by Atlassian JIRA (v6.2#6252)