[jira] [Commented] (KAFKA-1683) Implement a session concept in the socket server
[ https://issues.apache.org/jira/browse/KAFKA-1683?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14693676#comment-14693676 ] Eugene Miretsky commented on KAFKA-1683: My apologies, didn't word the question properly. I think that KAFKA-1686 solvers it - Kerberos support will allow authenticating as a specific user, and storing the user identity in a session for later authorization. Implement a session concept in the socket server -- Key: KAFKA-1683 URL: https://issues.apache.org/jira/browse/KAFKA-1683 Project: Kafka Issue Type: Sub-task Components: security Affects Versions: 0.9.0 Reporter: Jay Kreps Assignee: Gwen Shapira Fix For: 0.8.3 Attachments: KAFKA-1683.patch, KAFKA-1683.patch To implement authentication we need a way to keep track of some things between requests. The initial use for this would be remembering the authenticated user/principle info, but likely more uses would come up (for example we will also need to remember whether and which encryption or integrity measures are in place on the socket so we can wrap and unwrap writes and reads). I was thinking we could just add a Session object that might have a user field. The session object would need to get added to RequestChannel.Request so it is passed down to the API layer with each request. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Comment Edited] (KAFKA-1683) Implement a session concept in the socket server
[ https://issues.apache.org/jira/browse/KAFKA-1683?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14693676#comment-14693676 ] Eugene Miretsky edited comment on KAFKA-1683 at 8/12/15 3:36 PM: - My apologies, poorly worded question. I think that KAFKA-1686 solves it - Kerberos support will allow authenticating as a specific user, and storing the user identity in a session for later authorization. was (Author: emiretsk): My apologies, didn't word the question properly. I think that KAFKA-1686 solvers it - Kerberos support will allow authenticating as a specific user, and storing the user identity in a session for later authorization. Implement a session concept in the socket server -- Key: KAFKA-1683 URL: https://issues.apache.org/jira/browse/KAFKA-1683 Project: Kafka Issue Type: Sub-task Components: security Affects Versions: 0.9.0 Reporter: Jay Kreps Assignee: Gwen Shapira Fix For: 0.8.3 Attachments: KAFKA-1683.patch, KAFKA-1683.patch To implement authentication we need a way to keep track of some things between requests. The initial use for this would be remembering the authenticated user/principle info, but likely more uses would come up (for example we will also need to remember whether and which encryption or integrity measures are in place on the socket so we can wrap and unwrap writes and reads). I was thinking we could just add a Session object that might have a user field. The session object would need to get added to RequestChannel.Request so it is passed down to the API layer with each request. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Comment Edited] (KAFKA-1683) Implement a session concept in the socket server
[ https://issues.apache.org/jira/browse/KAFKA-1683?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14693676#comment-14693676 ] Eugene Miretsky edited comment on KAFKA-1683 at 8/12/15 3:39 PM: - My apologies, poorly worded question. Basically was asking where the user identity in the session will come from - 1-way SSL doesn't authenticate the client. I think that KAFKA-1686 will solve it - Kerberos support will allow authenticating as a specific user, and storing the user identity in a session for later authorization. was (Author: emiretsk): My apologies, poorly worded question. I think that KAFKA-1686 solves it - Kerberos support will allow authenticating as a specific user, and storing the user identity in a session for later authorization. Implement a session concept in the socket server -- Key: KAFKA-1683 URL: https://issues.apache.org/jira/browse/KAFKA-1683 Project: Kafka Issue Type: Sub-task Components: security Affects Versions: 0.9.0 Reporter: Jay Kreps Assignee: Gwen Shapira Fix For: 0.8.3 Attachments: KAFKA-1683.patch, KAFKA-1683.patch To implement authentication we need a way to keep track of some things between requests. The initial use for this would be remembering the authenticated user/principle info, but likely more uses would come up (for example we will also need to remember whether and which encryption or integrity measures are in place on the socket so we can wrap and unwrap writes and reads). I was thinking we could just add a Session object that might have a user field. The session object would need to get added to RequestChannel.Request so it is passed down to the API layer with each request. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Comment Edited] (KAFKA-1683) Implement a session concept in the socket server
[ https://issues.apache.org/jira/browse/KAFKA-1683?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14693676#comment-14693676 ] Eugene Miretsky edited comment on KAFKA-1683 at 8/12/15 3:40 PM: - My apologies, poorly worded question. Basically was asking where the user/client identity in the session will come from - 1-way SSL (KAFKA-1690) doesn't authenticate the client. I think that KAFKA-1686 will solve it - Kerberos support will allow authenticating as a specific user, and storing the user identity in a session for later authorization. was (Author: emiretsk): My apologies, poorly worded question. Basically was asking where the user identity in the session will come from - 1-way SSL doesn't authenticate the client. I think that KAFKA-1686 will solve it - Kerberos support will allow authenticating as a specific user, and storing the user identity in a session for later authorization. Implement a session concept in the socket server -- Key: KAFKA-1683 URL: https://issues.apache.org/jira/browse/KAFKA-1683 Project: Kafka Issue Type: Sub-task Components: security Affects Versions: 0.9.0 Reporter: Jay Kreps Assignee: Gwen Shapira Fix For: 0.8.3 Attachments: KAFKA-1683.patch, KAFKA-1683.patch To implement authentication we need a way to keep track of some things between requests. The initial use for this would be remembering the authenticated user/principle info, but likely more uses would come up (for example we will also need to remember whether and which encryption or integrity measures are in place on the socket so we can wrap and unwrap writes and reads). I was thinking we could just add a Session object that might have a user field. The session object would need to get added to RequestChannel.Request so it is passed down to the API layer with each request. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Work started] (KAFKA-1893) Allow regex subscriptions in the new consumer
[ https://issues.apache.org/jira/browse/KAFKA-1893?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Work on KAFKA-1893 started by Ashish K Singh. - Allow regex subscriptions in the new consumer - Key: KAFKA-1893 URL: https://issues.apache.org/jira/browse/KAFKA-1893 Project: Kafka Issue Type: Sub-task Components: consumer Reporter: Jay Kreps Assignee: Ashish K Singh Priority: Critical Fix For: 0.8.3 The consumer needs to handle subscribing to regular expressions. Presumably this would be done as a new api, {code} void subscribe(java.util.regex.Pattern pattern); {code} Some questions/thoughts to work out: - It should not be possible to mix pattern subscription with partition subscription. - Is it allowable to mix this with normal topic subscriptions? Logically this is okay but a bit complex to implement. - We need to ensure we regularly update the metadata and recheck our regexes against the metadata to update subscriptions for new topics that are created or old topics that are deleted. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Updated] (KAFKA-1893) Allow regex subscriptions in the new consumer
[ https://issues.apache.org/jira/browse/KAFKA-1893?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Ashish K Singh updated KAFKA-1893: -- Status: Patch Available (was: In Progress) Allow regex subscriptions in the new consumer - Key: KAFKA-1893 URL: https://issues.apache.org/jira/browse/KAFKA-1893 Project: Kafka Issue Type: Sub-task Components: consumer Reporter: Jay Kreps Assignee: Ashish K Singh Priority: Critical Fix For: 0.8.3 The consumer needs to handle subscribing to regular expressions. Presumably this would be done as a new api, {code} void subscribe(java.util.regex.Pattern pattern); {code} Some questions/thoughts to work out: - It should not be possible to mix pattern subscription with partition subscription. - Is it allowable to mix this with normal topic subscriptions? Logically this is okay but a bit complex to implement. - We need to ensure we regularly update the metadata and recheck our regexes against the metadata to update subscriptions for new topics that are created or old topics that are deleted. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[GitHub] kafka pull request: KAFKA-2429: Add annotations to mark classes as...
GitHub user ewencp opened a pull request: https://github.com/apache/kafka/pull/133 KAFKA-2429: Add annotations to mark classes as stable/unstable This also marks the consumer as unstable to show an example of using these annotations. You can merge this pull request into a Git repository by running: $ git pull https://github.com/ewencp/kafka stability-annotations Alternatively you can review and apply these changes as the patch at: https://github.com/apache/kafka/pull/133.patch To close this pull request, make a commit to your master/trunk branch with (at least) the following in the commit message: This closes #133 commit 09c15c37dcd128d608febbb9e578ef0ec85a471d Author: Ewen Cheslack-Postava m...@ewencp.org Date: 2015-08-12T21:04:01Z KAFKA-2429: Add annotations to mark classes as stable/unstable --- If your project is set up for it, you can reply to this email and have your reply appear on GitHub as well. If your project does not have this feature enabled and wishes so, or if the feature is enabled but not working, please contact infrastructure at infrastruct...@apache.org or file a JIRA ticket with INFRA. ---
[jira] [Commented] (KAFKA-2429) Add annotations to mark classes as stable/unstable
[ https://issues.apache.org/jira/browse/KAFKA-2429?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14694170#comment-14694170 ] ASF GitHub Bot commented on KAFKA-2429: --- GitHub user ewencp opened a pull request: https://github.com/apache/kafka/pull/133 KAFKA-2429: Add annotations to mark classes as stable/unstable This also marks the consumer as unstable to show an example of using these annotations. You can merge this pull request into a Git repository by running: $ git pull https://github.com/ewencp/kafka stability-annotations Alternatively you can review and apply these changes as the patch at: https://github.com/apache/kafka/pull/133.patch To close this pull request, make a commit to your master/trunk branch with (at least) the following in the commit message: This closes #133 commit 09c15c37dcd128d608febbb9e578ef0ec85a471d Author: Ewen Cheslack-Postava m...@ewencp.org Date: 2015-08-12T21:04:01Z KAFKA-2429: Add annotations to mark classes as stable/unstable Add annotations to mark classes as stable/unstable -- Key: KAFKA-2429 URL: https://issues.apache.org/jira/browse/KAFKA-2429 Project: Kafka Issue Type: Improvement Reporter: Ewen Cheslack-Postava Assignee: Ewen Cheslack-Postava We should have some annotations so that we can mark classes as public and stable vs. in development and unstable. This will help address two issues. First, we already get fairly regular emails on the mailing list about non-functioning code because we sometimes check in stubbed out code to get started on some new code. Sometimes that also makes it into a release (e.g. the stubbed out interface for the new consumer). We don't expect that code to work, but it's not obvious to users that it shouldn't. Second, we sometimes want to be able to check in imperfect draft code because it's new, expected to be unstable, and it helps with reviewing to be able to get something smaller checked in and then iterate on it. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
Re: Review Request 28096: Patch for KAFKA-313
On July 29, 2015, 6:35 p.m., Gwen Shapira wrote: core/src/main/scala/kafka/admin/ConsumerGroupCommand.scala, lines 237-242 https://reviews.apache.org/r/28096/diff/4/?file=991387#file991387line237 These look identical - copy/paste error? Ashish Singh wrote: Not really. There is some difference, None has %s, %s format, while CSV has %s,%s format. Gwen Shapira wrote: The KIP (https://cwiki.apache.org/confluence/pages/viewpage.action?pageId=56852556) doesn't mention NONE. Since CSV and NONE are so similar (just a matter of an extra space), does it make sense to just drop NONE? (which was my expectation, given the KIP) Gwen, I updated the patch to only have CSV and JSON as output-formats. - Ashish --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/28096/#review93489 --- On Aug. 12, 2015, 9:21 p.m., Ashish Singh wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/28096/ --- (Updated Aug. 12, 2015, 9:21 p.m.) Review request for kafka, Gwen Shapira, Jarek Cecho, and Joel Koshy. Bugs: KAFKA-313 https://issues.apache.org/jira/browse/KAFKA-313 Repository: kafka Description --- Remove NONE output-format KAFKA-313: Add JSON/CSV output and looping options to ConsumerGroupCommand Diffs - core/src/main/scala/kafka/admin/ConsumerGroupCommand.scala f23120ede5f9bf0cfaf795c65c9845f42d8784d0 Diff: https://reviews.apache.org/r/28096/diff/ Testing --- Ran ConsumerOffsetChecker with different combinations of --output.format and --loop options. Thanks, Ashish Singh
[jira] [Work started] (KAFKA-2429) Add annotations to mark classes as stable/unstable
[ https://issues.apache.org/jira/browse/KAFKA-2429?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Work on KAFKA-2429 started by Ewen Cheslack-Postava. Add annotations to mark classes as stable/unstable -- Key: KAFKA-2429 URL: https://issues.apache.org/jira/browse/KAFKA-2429 Project: Kafka Issue Type: Improvement Reporter: Ewen Cheslack-Postava Assignee: Ewen Cheslack-Postava We should have some annotations so that we can mark classes as public and stable vs. in development and unstable. This will help address two issues. First, we already get fairly regular emails on the mailing list about non-functioning code because we sometimes check in stubbed out code to get started on some new code. Sometimes that also makes it into a release (e.g. the stubbed out interface for the new consumer). We don't expect that code to work, but it's not obvious to users that it shouldn't. Second, we sometimes want to be able to check in imperfect draft code because it's new, expected to be unstable, and it helps with reviewing to be able to get something smaller checked in and then iterate on it. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
Re: Review Request 28096: Patch for KAFKA-313
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/28096/ --- (Updated Aug. 12, 2015, 9:21 p.m.) Review request for kafka, Gwen Shapira, Jarek Cecho, and Joel Koshy. Bugs: KAFKA-313 https://issues.apache.org/jira/browse/KAFKA-313 Repository: kafka Description (updated) --- Remove NONE output-format KAFKA-313: Add JSON/CSV output and looping options to ConsumerGroupCommand Diffs (updated) - core/src/main/scala/kafka/admin/ConsumerGroupCommand.scala f23120ede5f9bf0cfaf795c65c9845f42d8784d0 Diff: https://reviews.apache.org/r/28096/diff/ Testing --- Ran ConsumerOffsetChecker with different combinations of --output.format and --loop options. Thanks, Ashish Singh
[jira] [Commented] (KAFKA-313) Add JSON/CSV output and looping options to ConsumerGroupCommand
[ https://issues.apache.org/jira/browse/KAFKA-313?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14694190#comment-14694190 ] Ashish K Singh commented on KAFKA-313: -- Updated reviewboard https://reviews.apache.org/r/28096/ against branch trunk Add JSON/CSV output and looping options to ConsumerGroupCommand --- Key: KAFKA-313 URL: https://issues.apache.org/jira/browse/KAFKA-313 Project: Kafka Issue Type: Improvement Reporter: Dave DeMaagd Assignee: Ashish K Singh Priority: Minor Labels: newbie, patch Fix For: 0.8.3 Attachments: KAFKA-313-2012032200.diff, KAFKA-313.1.patch, KAFKA-313.patch, KAFKA-313_2015-02-23_18:11:32.patch, KAFKA-313_2015-06-24_11:14:24.patch, KAFKA-313_2015-08-05_15:37:32.patch, KAFKA-313_2015-08-05_15:43:00.patch, KAFKA-313_2015-08-10_12:58:38.patch, KAFKA-313_2015-08-12_14:21:32.patch Adds: * '--loop N' - causes the program to loop forever, sleeping for up to N seconds between loops (loop time minus collection time, unless that's less than 0, at which point it will just run again immediately) * '--asjson' - display as a JSON string instead of the more human readable output format. Neither of the above depend on each other (you can loop in the human readable output, or do a single shot execution with JSON output). Existing behavior/output maintained if neither of the above are used. Diff Attached. Impacted files: core/src/main/scala/kafka/admin/ConsumerGroupCommand.scala -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Updated] (KAFKA-2429) Add annotations to mark classes as stable/unstable
[ https://issues.apache.org/jira/browse/KAFKA-2429?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Gwen Shapira updated KAFKA-2429: Resolution: Fixed Fix Version/s: 0.8.3 Status: Resolved (was: Patch Available) Issue resolved by pull request 133 [https://github.com/apache/kafka/pull/133] Add annotations to mark classes as stable/unstable -- Key: KAFKA-2429 URL: https://issues.apache.org/jira/browse/KAFKA-2429 Project: Kafka Issue Type: Improvement Reporter: Ewen Cheslack-Postava Assignee: Ewen Cheslack-Postava Fix For: 0.8.3 We should have some annotations so that we can mark classes as public and stable vs. in development and unstable. This will help address two issues. First, we already get fairly regular emails on the mailing list about non-functioning code because we sometimes check in stubbed out code to get started on some new code. Sometimes that also makes it into a release (e.g. the stubbed out interface for the new consumer). We don't expect that code to work, but it's not obvious to users that it shouldn't. Second, we sometimes want to be able to check in imperfect draft code because it's new, expected to be unstable, and it helps with reviewing to be able to get something smaller checked in and then iterate on it. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Commented] (KAFKA-2429) Add annotations to mark classes as stable/unstable
[ https://issues.apache.org/jira/browse/KAFKA-2429?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14694249#comment-14694249 ] ASF GitHub Bot commented on KAFKA-2429: --- Github user asfgit closed the pull request at: https://github.com/apache/kafka/pull/133 Add annotations to mark classes as stable/unstable -- Key: KAFKA-2429 URL: https://issues.apache.org/jira/browse/KAFKA-2429 Project: Kafka Issue Type: Improvement Reporter: Ewen Cheslack-Postava Assignee: Ewen Cheslack-Postava Fix For: 0.8.3 We should have some annotations so that we can mark classes as public and stable vs. in development and unstable. This will help address two issues. First, we already get fairly regular emails on the mailing list about non-functioning code because we sometimes check in stubbed out code to get started on some new code. Sometimes that also makes it into a release (e.g. the stubbed out interface for the new consumer). We don't expect that code to work, but it's not obvious to users that it shouldn't. Second, we sometimes want to be able to check in imperfect draft code because it's new, expected to be unstable, and it helps with reviewing to be able to get something smaller checked in and then iterate on it. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Commented] (KAFKA-2425) Migrate website from SVN to Git
[ https://issues.apache.org/jira/browse/KAFKA-2425?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14694393#comment-14694393 ] Ismael Juma commented on KAFKA-2425: Requiring a different branch is annoying. However, I think we can still push code and doc together [~gwenshap]. It just means that the website won't be updated until we merge the changes to the `asf-site` branch, right? Or am I missing something? Maybe this extra step is annoying enough that having all in a single repo is no longer worth it. Not sure. Migrate website from SVN to Git Key: KAFKA-2425 URL: https://issues.apache.org/jira/browse/KAFKA-2425 Project: Kafka Issue Type: Improvement Reporter: Ismael Juma The preference is to share the same Git repo for the code and website as per discussion in the mailing list: http://search-hadoop.com/m/uyzND1Dux842dm7vg2 Useful reference: https://blogs.apache.org/infra/entry/git_based_websites_available -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Updated] (KAFKA-2300) Error in controller log when broker tries to rejoin cluster
[ https://issues.apache.org/jira/browse/KAFKA-2300?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Guozhang Wang updated KAFKA-2300: - Resolution: Fixed Status: Resolved (was: Patch Available) Issue resolved by pull request 102 [https://github.com/apache/kafka/pull/102] Error in controller log when broker tries to rejoin cluster --- Key: KAFKA-2300 URL: https://issues.apache.org/jira/browse/KAFKA-2300 Project: Kafka Issue Type: Bug Affects Versions: 0.8.2.1 Reporter: Johnny Brown Assignee: Flavio Junqueira Fix For: 0.8.3 Attachments: KAFKA-2300-controller-logs.tar.gz, KAFKA-2300-repro.patch, KAFKA-2300.patch, KAFKA-2300.patch Hello Kafka folks, We are having an issue where a broker attempts to join the cluster after being restarted, but is never added to the ISR for its assigned partitions. This is a three-node cluster, and the controller is broker 2. When broker 1 starts, we see the following message in broker 2's controller.log. {{ [2015-06-23 13:57:16,535] ERROR [BrokerChangeListener on Controller 2]: Error while handling broker changes (kafka.controller.ReplicaStateMachine$BrokerChangeListener) java.lang.IllegalStateException: Controller to broker state change requests batch is not empty while creating a new one. Some UpdateMetadata state changes Map(2 - Map([prod-sver-end,1] - (LeaderAndIsrInfo:(Leader:-2,ISR:1,LeaderEpoch:0,ControllerEpoch:165),ReplicationFactor:1),AllReplicas:1)), 1 - Map([prod-sver-end,1] - (LeaderAndIsrInfo:(Leader:-2,ISR:1,LeaderEpoch:0,ControllerEpoch:165),ReplicationFactor:1),AllReplicas:1)), 3 - Map([prod-sver-end,1] - (LeaderAndIsrInfo:(Leader:-2,ISR:1,LeaderEpoch:0,ControllerEpoch:165),ReplicationFactor:1),AllReplicas:1))) might be lost at kafka.controller.ControllerBrokerRequestBatch.newBatch(ControllerChannelManager.scala:202) at kafka.controller.KafkaController.sendUpdateMetadataRequest(KafkaController.scala:974) at kafka.controller.KafkaController.onBrokerStartup(KafkaController.scala:399) at kafka.controller.ReplicaStateMachine$BrokerChangeListener$$anonfun$handleChildChange$1$$anonfun$apply$mcV$sp$1.apply$mcV$sp(ReplicaStateMachine.scala:371) at kafka.controller.ReplicaStateMachine$BrokerChangeListener$$anonfun$handleChildChange$1$$anonfun$apply$mcV$sp$1.apply(ReplicaStateMachine.scala:359) at kafka.controller.ReplicaStateMachine$BrokerChangeListener$$anonfun$handleChildChange$1$$anonfun$apply$mcV$sp$1.apply(ReplicaStateMachine.scala:359) at kafka.metrics.KafkaTimer.time(KafkaTimer.scala:33) at kafka.controller.ReplicaStateMachine$BrokerChangeListener$$anonfun$handleChildChange$1.apply$mcV$sp(ReplicaStateMachine.scala:358) at kafka.controller.ReplicaStateMachine$BrokerChangeListener$$anonfun$handleChildChange$1.apply(ReplicaStateMachine.scala:357) at kafka.controller.ReplicaStateMachine$BrokerChangeListener$$anonfun$handleChildChange$1.apply(ReplicaStateMachine.scala:357) at kafka.utils.Utils$.inLock(Utils.scala:535) at kafka.controller.ReplicaStateMachine$BrokerChangeListener.handleChildChange(ReplicaStateMachine.scala:356) at org.I0Itec.zkclient.ZkClient$7.run(ZkClient.java:568) at org.I0Itec.zkclient.ZkEventThread.run(ZkEventThread.java:71) }} {{prod-sver-end}} is a topic we previously deleted. It seems some remnant of it persists in the controller's memory, causing an exception which interrupts the state change triggered by the broker startup. Has anyone seen something like this? Any idea what's happening here? Any information would be greatly appreciated. Thanks, Johnny -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Commented] (KAFKA-2300) Error in controller log when broker tries to rejoin cluster
[ https://issues.apache.org/jira/browse/KAFKA-2300?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14694208#comment-14694208 ] ASF GitHub Bot commented on KAFKA-2300: --- Github user asfgit closed the pull request at: https://github.com/apache/kafka/pull/102 Error in controller log when broker tries to rejoin cluster --- Key: KAFKA-2300 URL: https://issues.apache.org/jira/browse/KAFKA-2300 Project: Kafka Issue Type: Bug Affects Versions: 0.8.2.1 Reporter: Johnny Brown Assignee: Flavio Junqueira Fix For: 0.8.3 Attachments: KAFKA-2300-controller-logs.tar.gz, KAFKA-2300-repro.patch, KAFKA-2300.patch, KAFKA-2300.patch Hello Kafka folks, We are having an issue where a broker attempts to join the cluster after being restarted, but is never added to the ISR for its assigned partitions. This is a three-node cluster, and the controller is broker 2. When broker 1 starts, we see the following message in broker 2's controller.log. {{ [2015-06-23 13:57:16,535] ERROR [BrokerChangeListener on Controller 2]: Error while handling broker changes (kafka.controller.ReplicaStateMachine$BrokerChangeListener) java.lang.IllegalStateException: Controller to broker state change requests batch is not empty while creating a new one. Some UpdateMetadata state changes Map(2 - Map([prod-sver-end,1] - (LeaderAndIsrInfo:(Leader:-2,ISR:1,LeaderEpoch:0,ControllerEpoch:165),ReplicationFactor:1),AllReplicas:1)), 1 - Map([prod-sver-end,1] - (LeaderAndIsrInfo:(Leader:-2,ISR:1,LeaderEpoch:0,ControllerEpoch:165),ReplicationFactor:1),AllReplicas:1)), 3 - Map([prod-sver-end,1] - (LeaderAndIsrInfo:(Leader:-2,ISR:1,LeaderEpoch:0,ControllerEpoch:165),ReplicationFactor:1),AllReplicas:1))) might be lost at kafka.controller.ControllerBrokerRequestBatch.newBatch(ControllerChannelManager.scala:202) at kafka.controller.KafkaController.sendUpdateMetadataRequest(KafkaController.scala:974) at kafka.controller.KafkaController.onBrokerStartup(KafkaController.scala:399) at kafka.controller.ReplicaStateMachine$BrokerChangeListener$$anonfun$handleChildChange$1$$anonfun$apply$mcV$sp$1.apply$mcV$sp(ReplicaStateMachine.scala:371) at kafka.controller.ReplicaStateMachine$BrokerChangeListener$$anonfun$handleChildChange$1$$anonfun$apply$mcV$sp$1.apply(ReplicaStateMachine.scala:359) at kafka.controller.ReplicaStateMachine$BrokerChangeListener$$anonfun$handleChildChange$1$$anonfun$apply$mcV$sp$1.apply(ReplicaStateMachine.scala:359) at kafka.metrics.KafkaTimer.time(KafkaTimer.scala:33) at kafka.controller.ReplicaStateMachine$BrokerChangeListener$$anonfun$handleChildChange$1.apply$mcV$sp(ReplicaStateMachine.scala:358) at kafka.controller.ReplicaStateMachine$BrokerChangeListener$$anonfun$handleChildChange$1.apply(ReplicaStateMachine.scala:357) at kafka.controller.ReplicaStateMachine$BrokerChangeListener$$anonfun$handleChildChange$1.apply(ReplicaStateMachine.scala:357) at kafka.utils.Utils$.inLock(Utils.scala:535) at kafka.controller.ReplicaStateMachine$BrokerChangeListener.handleChildChange(ReplicaStateMachine.scala:356) at org.I0Itec.zkclient.ZkClient$7.run(ZkClient.java:568) at org.I0Itec.zkclient.ZkEventThread.run(ZkEventThread.java:71) }} {{prod-sver-end}} is a topic we previously deleted. It seems some remnant of it persists in the controller's memory, causing an exception which interrupts the state change triggered by the broker startup. Has anyone seen something like this? Any idea what's happening here? Any information would be greatly appreciated. Thanks, Johnny -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[GitHub] kafka pull request: KAFKA-2300: Error in controller log when broke...
Github user asfgit closed the pull request at: https://github.com/apache/kafka/pull/102 --- If your project is set up for it, you can reply to this email and have your reply appear on GitHub as well. If your project does not have this feature enabled and wishes so, or if the feature is enabled but not working, please contact infrastructure at infrastruct...@apache.org or file a JIRA ticket with INFRA. ---
[jira] [Updated] (KAFKA-2429) Add annotations to mark classes as stable/unstable
[ https://issues.apache.org/jira/browse/KAFKA-2429?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Ewen Cheslack-Postava updated KAFKA-2429: - Status: Patch Available (was: In Progress) Add annotations to mark classes as stable/unstable -- Key: KAFKA-2429 URL: https://issues.apache.org/jira/browse/KAFKA-2429 Project: Kafka Issue Type: Improvement Reporter: Ewen Cheslack-Postava Assignee: Ewen Cheslack-Postava We should have some annotations so that we can mark classes as public and stable vs. in development and unstable. This will help address two issues. First, we already get fairly regular emails on the mailing list about non-functioning code because we sometimes check in stubbed out code to get started on some new code. Sometimes that also makes it into a release (e.g. the stubbed out interface for the new consumer). We don't expect that code to work, but it's not obvious to users that it shouldn't. Second, we sometimes want to be able to check in imperfect draft code because it's new, expected to be unstable, and it helps with reviewing to be able to get something smaller checked in and then iterate on it. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[GitHub] kafka pull request: MINOR: Fix hard coded strings in ProduceRespon...
Github user asfgit closed the pull request at: https://github.com/apache/kafka/pull/131 --- If your project is set up for it, you can reply to this email and have your reply appear on GitHub as well. If your project does not have this feature enabled and wishes so, or if the feature is enabled but not working, please contact infrastructure at infrastruct...@apache.org or file a JIRA ticket with INFRA. ---
[jira] [Commented] (KAFKA-2367) Add Copycat runtime data API
[ https://issues.apache.org/jira/browse/KAFKA-2367?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14694383#comment-14694383 ] Ismael Juma commented on KAFKA-2367: I think using Avro (as suggested by [~gwenshap]) seems preferable unless we think the field #s are important. Without field #s, it seems to me that the second class argument for other formats is not particularly important (i.e. they need to be converted either way and both our own format and Avro would be similar). The fact that Avro has even better support is not a bad thing IMO (only one format can be native after all). It's a bit like slf4j and logback, the latter implements the interface directly and doesn't need adapters. Add Copycat runtime data API Key: KAFKA-2367 URL: https://issues.apache.org/jira/browse/KAFKA-2367 Project: Kafka Issue Type: Sub-task Components: copycat Reporter: Ewen Cheslack-Postava Assignee: Ewen Cheslack-Postava Fix For: 0.8.3 Design the API used for runtime data in Copycat. This API is used to construct schemas and records that Copycat processes. This needs to be a fairly general data model (think Avro, JSON, Protobufs, Thrift) in order to support complex, varied data types that may be input from/output to many data systems. This should issue should also address the serialization interfaces used within Copycat, which translate the runtime data into serialized byte[] form. It is important that these be considered together because the data format can be used in multiple ways (records, partition IDs, partition offsets), so it and the corresponding serializers must be sufficient for all these use cases. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Updated] (KAFKA-313) Add JSON/CSV output and looping options to ConsumerGroupCommand
[ https://issues.apache.org/jira/browse/KAFKA-313?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Ashish K Singh updated KAFKA-313: - Attachment: KAFKA-313_2015-08-12_14:21:32.patch Add JSON/CSV output and looping options to ConsumerGroupCommand --- Key: KAFKA-313 URL: https://issues.apache.org/jira/browse/KAFKA-313 Project: Kafka Issue Type: Improvement Reporter: Dave DeMaagd Assignee: Ashish K Singh Priority: Minor Labels: newbie, patch Fix For: 0.8.3 Attachments: KAFKA-313-2012032200.diff, KAFKA-313.1.patch, KAFKA-313.patch, KAFKA-313_2015-02-23_18:11:32.patch, KAFKA-313_2015-06-24_11:14:24.patch, KAFKA-313_2015-08-05_15:37:32.patch, KAFKA-313_2015-08-05_15:43:00.patch, KAFKA-313_2015-08-10_12:58:38.patch, KAFKA-313_2015-08-12_14:21:32.patch Adds: * '--loop N' - causes the program to loop forever, sleeping for up to N seconds between loops (loop time minus collection time, unless that's less than 0, at which point it will just run again immediately) * '--asjson' - display as a JSON string instead of the more human readable output format. Neither of the above depend on each other (you can loop in the human readable output, or do a single shot execution with JSON output). Existing behavior/output maintained if neither of the above are used. Diff Attached. Impacted files: core/src/main/scala/kafka/admin/ConsumerGroupCommand.scala -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Updated] (KAFKA-313) Add JSON/CSV output and looping options to ConsumerGroupCommand
[ https://issues.apache.org/jira/browse/KAFKA-313?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Ashish K Singh updated KAFKA-313: - Status: Patch Available (was: In Progress) Add JSON/CSV output and looping options to ConsumerGroupCommand --- Key: KAFKA-313 URL: https://issues.apache.org/jira/browse/KAFKA-313 Project: Kafka Issue Type: Improvement Reporter: Dave DeMaagd Assignee: Ashish K Singh Priority: Minor Labels: newbie, patch Fix For: 0.8.3 Attachments: KAFKA-313-2012032200.diff, KAFKA-313.1.patch, KAFKA-313.patch, KAFKA-313_2015-02-23_18:11:32.patch, KAFKA-313_2015-06-24_11:14:24.patch, KAFKA-313_2015-08-05_15:37:32.patch, KAFKA-313_2015-08-05_15:43:00.patch, KAFKA-313_2015-08-10_12:58:38.patch, KAFKA-313_2015-08-12_14:21:32.patch Adds: * '--loop N' - causes the program to loop forever, sleeping for up to N seconds between loops (loop time minus collection time, unless that's less than 0, at which point it will just run again immediately) * '--asjson' - display as a JSON string instead of the more human readable output format. Neither of the above depend on each other (you can loop in the human readable output, or do a single shot execution with JSON output). Existing behavior/output maintained if neither of the above are used. Diff Attached. Impacted files: core/src/main/scala/kafka/admin/ConsumerGroupCommand.scala -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[GitHub] kafka pull request: KAFKA-2429: Add annotations to mark classes as...
Github user asfgit closed the pull request at: https://github.com/apache/kafka/pull/133 --- If your project is set up for it, you can reply to this email and have your reply appear on GitHub as well. If your project does not have this feature enabled and wishes so, or if the feature is enabled but not working, please contact infrastructure at infrastruct...@apache.org or file a JIRA ticket with INFRA. ---
NEW: API Stability annotations!
Hi Team Kafka, Ewen just added stability annotations to Apache Kafka (KAFKA-2429). In the same PR, we marked the new Consumer API as unstable since we are still actively iterating on them. The goal is to mark them as evolving before the next release and after one release to validate them, we will mark them as stable. When adding new public APIs, we encourage you to think of their stage of development and annotate correctly. The usage is: - unstable - can change at any time - evolving - can break compatibility at minor releases (i.e. 0.9 may be incompatible with 0.8) - stable - will only break compatibility at major releases (1.0, 2.0, etc) Enjoy! Gwen
[jira] [Created] (KAFKA-2429) Add annotations to mark classes as stable/unstable
Ewen Cheslack-Postava created KAFKA-2429: Summary: Add annotations to mark classes as stable/unstable Key: KAFKA-2429 URL: https://issues.apache.org/jira/browse/KAFKA-2429 Project: Kafka Issue Type: Improvement Reporter: Ewen Cheslack-Postava Assignee: Ewen Cheslack-Postava We should have some annotations so that we can mark classes as public and stable vs. in development and unstable. This will help address two issues. First, we already get fairly regular emails on the mailing list about non-functioning code because we sometimes check in stubbed out code to get started on some new code. Sometimes that also makes it into a release (e.g. the stubbed out interface for the new consumer). We don't expect that code to work, but it's not obvious to users that it shouldn't. Second, we sometimes want to be able to check in imperfect draft code because it's new, expected to be unstable, and it helps with reviewing to be able to get something smaller checked in and then iterate on it. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
Re: KIP Meeting Notes 08/11/2015
Yeah. Jun used to send out table-summary of backlogs with reviews before KIP meeting, we can continue to do that. Also we used to have a dashboard for replication development tracking, Neha/Jun/Joel do you remember how to setup sth. similar? Guozhang On Tue, Aug 11, 2015 at 10:31 PM, Jiangjie Qin j...@linkedin.com.invalid wrote: Hey Guozhang, Will it be a little bit hard to keep the volunteer list up to date? Personally I would prefer to have a summery e-mail automatically sent to kafka-dev list every day for tickets with patches submitted in recent 7 days. The email can also include the reviewer for the ticket. And people can just take a look a the patch if it is not assigned to anyone. Similarly we can also list the tickets that has been open for some time but haven't been updated or closed. If getting email everyday is too much we can also do it weekly, although I think people won't complain for one more email given there are already tons of emails every day :) Thanks, Jiangjie (Becket) QIn On Tue, Aug 11, 2015 at 3:47 PM, Guozhang Wang wangg...@gmail.com wrote: Good question. I can personally think of pros and cons of having a volunteer list, most of them are pros but one con is that the list will never be comprehensive and in that sense sort of discouraging people to assign themselves as the reviewer. Without such a list, contributors would most likely assign reviewers to who they saw to have been a reviewer before or who they know of (i.e. a committer most of times). But we could try to encourage people re-assign review roles to who they think would be comfortable to do so (maybe they have contributed multiple patches on that module, or they have participated discussions in that topic, or they are known to have the background, etc), while at the same time encourage people to (re-)assign reviewer to themselves, and hope that over time more people to be observed as the reviewers to go to. This may also help the community to grow committers. Thoughts? Guozhang On Tue, Aug 11, 2015 at 1:50 PM, Grant Henke ghe...@cloudera.com wrote: 2. Encourage contributors to set the reviewer field when change JIRA status to patch available, and encourage volunteers assigning themselves to reviewers for pending tickets. Is there somewhere that describes who to pick as a reviewer based on the patch? Would it be worth listing volunteer reviews in a similar location? On Tue, Aug 11, 2015 at 2:14 PM, Guozhang Wang wangg...@gmail.com wrote: First of all, WebEx seems working! And we will upload the recorded video later. Quick summary: KIP-26: RP-99 (https://github.com/apache/kafka/pull/99) pending for reviews. KIP-28: RP-130 (https://github.com/apache/kafka/pull/130) looking for feedbacks on: 1. API design (see o.k.a.stream.examples). 2. Architecture design (see KIP wiki page) 3. Packaging options. KIP-29: we will do a quick fix for unblocking production issues with hard-coded interval values, while at the same time keep the KIP open for further discussions about end state configurations. KIP-4: KAFKA-1695 / 2210 pending for reviews. Review Backlog Management: 1. Remind people to change JIRA status as patch available when they contribute the patch, and change the status back to in progress after it is reviewed, as indicated in: https://cwiki.apache.org/confluence/display/KAFKA/Contributing+Code+Changes 2. Encourage contributors to set the reviewer field when change JIRA status to patch available, and encourage volunteers assigning themselves to reviewers for pending tickets. -- Guozhang -- Grant Henke Software Engineer | Cloudera gr...@cloudera.com | twitter.com/gchenke | linkedin.com/in/granthenke -- -- Guozhang -- -- Guozhang
[jira] [Commented] (KAFKA-2367) Add Copycat runtime data API
[ https://issues.apache.org/jira/browse/KAFKA-2367?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14694407#comment-14694407 ] Gwen Shapira commented on KAFKA-2367: - I'm a strong +1 on using Avro instead of maintaining our own clone of selected parts. Main reason is that in Sqoop2 we maintained a clone of selected parts and we spent way too much time fixing issues in that layer and maintaining it. Avro is well... Avro. Its been used and tested quite a bit. I agree that Avro is not perfect, however, we can work with the Avro community to resolve particularly painful issues. Add Copycat runtime data API Key: KAFKA-2367 URL: https://issues.apache.org/jira/browse/KAFKA-2367 Project: Kafka Issue Type: Sub-task Components: copycat Reporter: Ewen Cheslack-Postava Assignee: Ewen Cheslack-Postava Fix For: 0.8.3 Design the API used for runtime data in Copycat. This API is used to construct schemas and records that Copycat processes. This needs to be a fairly general data model (think Avro, JSON, Protobufs, Thrift) in order to support complex, varied data types that may be input from/output to many data systems. This should issue should also address the serialization interfaces used within Copycat, which translate the runtime data into serialized byte[] form. It is important that these be considered together because the data format can be used in multiple ways (records, partition IDs, partition offsets), so it and the corresponding serializers must be sufficient for all these use cases. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
Re: NEW: API Stability annotations!
This is a great idea! I do question the classifications a bit. @stable seams unreasonable as Kafka has not has a major release yet. I have typically thought of Kafka releases as 0.{major}.{minor}.{maintenance}. For instance 0.8.3 coming up has a significant amount of change. Though Kafka's release cycle could be changing and I may not understand the cycle/classification correctly. I am thinking something along the lines of: - unstable - can change at any time (i.e maintenance releases) - evolving - can break compatibility at minor releases (i.e. 0.8.3 may be incompatible with 0.8.2) - stable - will only break compatibility at major releases (0.8, 0.9, etc) Regardless the choice of the contract, I do suggest we document this. On Wed, Aug 12, 2015 at 5:05 PM, Gwen Shapira g...@confluent.io wrote: Hi Team Kafka, Ewen just added stability annotations to Apache Kafka (KAFKA-2429). In the same PR, we marked the new Consumer API as unstable since we are still actively iterating on them. The goal is to mark them as evolving before the next release and after one release to validate them, we will mark them as stable. When adding new public APIs, we encourage you to think of their stage of development and annotate correctly. The usage is: - unstable - can change at any time - evolving - can break compatibility at minor releases (i.e. 0.9 may be incompatible with 0.8) - stable - will only break compatibility at major releases (1.0, 2.0, etc) Enjoy! Gwen -- Grant Henke Software Engineer | Cloudera gr...@cloudera.com | twitter.com/gchenke | linkedin.com/in/granthenke
Re: NEW: API Stability annotations!
Yeah, I agree that your description matches reality better :) By document do you have any suggestions where? The interface itself has documentation. Were you thinking Wiki? On Wed, Aug 12, 2015 at 3:48 PM, Grant Henke ghe...@cloudera.com wrote: This is a great idea! I do question the classifications a bit. @stable seams unreasonable as Kafka has not has a major release yet. I have typically thought of Kafka releases as 0.{major}.{minor}.{maintenance}. For instance 0.8.3 coming up has a significant amount of change. Though Kafka's release cycle could be changing and I may not understand the cycle/classification correctly. I am thinking something along the lines of: - unstable - can change at any time (i.e maintenance releases) - evolving - can break compatibility at minor releases (i.e. 0.8.3 may be incompatible with 0.8.2) - stable - will only break compatibility at major releases (0.8, 0.9, etc) Regardless the choice of the contract, I do suggest we document this. On Wed, Aug 12, 2015 at 5:05 PM, Gwen Shapira g...@confluent.io wrote: Hi Team Kafka, Ewen just added stability annotations to Apache Kafka (KAFKA-2429). In the same PR, we marked the new Consumer API as unstable since we are still actively iterating on them. The goal is to mark them as evolving before the next release and after one release to validate them, we will mark them as stable. When adding new public APIs, we encourage you to think of their stage of development and annotate correctly. The usage is: - unstable - can change at any time - evolving - can break compatibility at minor releases (i.e. 0.9 may be incompatible with 0.8) - stable - will only break compatibility at major releases (1.0, 2.0, etc) Enjoy! Gwen -- Grant Henke Software Engineer | Cloudera gr...@cloudera.com | twitter.com/gchenke | linkedin.com/in/granthenke
Re: Review Request 33049: Patch for KAFKA-2084
On Aug. 6, 2015, 4:17 p.m., Jun Rao wrote: A few more comments. We need to be careful with sensors at the client-id level. Clients can come and go (e.g. console consumer). We probably don't want to hold sensors that are not longer actively used since it takes memory. So, we will need some way of removing inactive sensors. Not sure if we should add this at the metric level or at the quota level. Jun Rao wrote: Did you address the comment on removing inactive sensors? Aditya Auradkar wrote: Ah, I missed this comment. Good point.. we should be removing these sensor objects. I think we should handle this in the Metrics library itself.. it would be nice to support sensors that can be garbage collected after a certain period of inactivity (if the sensor is marked as eligible for removal). The new metrics library does not support removal of sensors right now so I filed a ticket as followup since it might need a bit more discussion: https://issues.apache.org/jira/browse/KAFKA-2419 @Jun - good point. Aditya, minor feedback on that ticket: it may be better to not make it time-based (config driven) but proactively remove sensors when required. E.g., when a client closes a connection (for client-id sensors) or topics get deleted (for topic sensors) and so on. - Joel --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/33049/#review94412 --- On Aug. 12, 2015, 7:09 p.m., Aditya Auradkar wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/33049/ --- (Updated Aug. 12, 2015, 7:09 p.m.) Review request for kafka, Joel Koshy and Jun Rao. Bugs: KAFKA-2084 https://issues.apache.org/jira/browse/KAFKA-2084 Repository: kafka Description --- Updated patch for quotas. This patch does the following: 1. Add per-client metrics for both producer and consumers 2. Add configuration for quotas 3. Compute delay times in the metrics package and return the delay times in QuotaViolationException 4. Add a DelayQueue in KafkaApi's that can be used to throttle any type of request. Implemented request throttling for produce and fetch requests. 5. Added unit and integration test cases for both producer and consumer 6. This doesn't include a system test. There is a separate ticket for that 7. Fixed KAFKA-2191 - (Included fix from : https://reviews.apache.org/r/34418/ ) Addressed comments from Joel and Jun Diffs - clients/src/main/java/org/apache/kafka/common/metrics/Quota.java d82bb0c055e631425bc1ebbc7d387baac76aeeaa clients/src/main/java/org/apache/kafka/common/metrics/QuotaViolationException.java a451e5385c9eca76b38b425e8ac856b2715fcffe clients/src/main/java/org/apache/kafka/common/metrics/Sensor.java ca823fd4639523018311b814fde69b6177e73b97 clients/src/main/java/org/apache/kafka/common/metrics/stats/Rate.java 98429da34418f7f1deba1b5e44e2e6025212edb3 clients/src/test/java/org/apache/kafka/common/metrics/MetricsTest.java 544e120594de78c43581a980b1e4087b4fb98ccb clients/src/test/java/org/apache/kafka/common/utils/MockTime.java core/src/main/scala/kafka/server/ClientQuotaManager.scala PRE-CREATION core/src/main/scala/kafka/server/KafkaApis.scala 7ea509c2c41acc00430c74e025e069a833aac4e7 core/src/main/scala/kafka/server/KafkaConfig.scala dbe170f87331f43e2dc30165080d2cb7dfe5fdbf core/src/main/scala/kafka/server/KafkaServer.scala 84d4730ac634f9a5bf12a656e422fea03ad72da8 core/src/main/scala/kafka/server/ReplicaManager.scala 795220e7f63d163be90738b4c1a39687b44c1395 core/src/main/scala/kafka/server/ThrottledResponse.scala PRE-CREATION core/src/main/scala/kafka/utils/ShutdownableThread.scala fc226c863095b7761290292cd8755cd7ad0f155c core/src/test/scala/integration/kafka/api/QuotasTest.scala PRE-CREATION core/src/test/scala/unit/kafka/server/ClientQuotaManagerTest.scala PRE-CREATION core/src/test/scala/unit/kafka/server/KafkaConfigTest.scala f32d206d3f52f3f9f4d649c213edd7058f4b6150 core/src/test/scala/unit/kafka/server/ThrottledResponseExpirationTest.scala PRE-CREATION Diff: https://reviews.apache.org/r/33049/diff/ Testing --- Thanks, Aditya Auradkar
[jira] [Resolved] (KAFKA-1782) Junit3 Misusage
[ https://issues.apache.org/jira/browse/KAFKA-1782?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Guozhang Wang resolved KAFKA-1782. -- Resolution: Fixed Issue resolved by pull request 135 [https://github.com/apache/kafka/pull/135] Junit3 Misusage --- Key: KAFKA-1782 URL: https://issues.apache.org/jira/browse/KAFKA-1782 Project: Kafka Issue Type: Bug Reporter: Guozhang Wang Assignee: Alexander Pakulov Labels: newbie Fix For: 0.8.3 Attachments: KAFKA-1782.patch, KAFKA-1782.patch, KAFKA-1782_2015-06-18_11:52:49.patch, KAFKA-1782_2015-07-15_16:57:44.patch, KAFKA-1782_2015-07-16_11:50:05.patch, KAFKA-1782_2015-07-16_11:56:11.patch This is found while I was working on KAFKA-1580: in many of our cases where we explicitly extend from junit3suite (e.g. ProducerFailureHandlingTest), we are actually misusing a bunch of features that only exist in Junit4, such as (expected=classOf). For example, the following code {code} import org.scalatest.junit.JUnit3Suite import org.junit.Test import java.io.IOException class MiscTest extends JUnit3Suite { @Test (expected = classOf[IOException]) def testSendOffset() { } } {code} will actually pass even though IOException was not thrown since this annotation is not supported in Junit3. Whereas {code} import org.junit._ import java.io.IOException class MiscTest extends JUnit3Suite { @Test (expected = classOf[IOException]) def testSendOffset() { } } {code} or {code} import org.scalatest.junit.JUnitSuite import org.junit._ import java.io.IOException class MiscTest extends JUnit3Suite { @Test (expected = classOf[IOException]) def testSendOffset() { } } {code} or {code} import org.junit._ import java.io.IOException class MiscTest { @Test (expected = classOf[IOException]) def testSendOffset() { } } {code} will fail. I would propose to not rely on Junit annotations other than @Test itself but use scala unit test annotations instead, for example: {code} import org.junit._ import java.io.IOException class MiscTest { @Test def testSendOffset() { intercept[IOException] { //nothing } } } {code} will fail with a clearer stacktrace. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[GitHub] kafka pull request: KAFKA-2389: remove commit type from new consum...
GitHub user becketqin opened a pull request: https://github.com/apache/kafka/pull/134 KAFKA-2389: remove commit type from new consumer. A shot to remove commit type from new consumer. The coordinator constructor takes a default offset commit callback mainly for testing purpose. You can merge this pull request into a Git repository by running: $ git pull https://github.com/becketqin/kafka KAFKA-2389 Alternatively you can review and apply these changes as the patch at: https://github.com/apache/kafka/pull/134.patch To close this pull request, make a commit to your master/trunk branch with (at least) the following in the commit message: This closes #134 commit 5f332efa6c690fad730278283d8c419c6a223a8e Author: Jiangjie Qin becket@gmail.com Date: 2015-08-11T19:41:15Z KAFKA-2389: Remove commit type from commit() --- If your project is set up for it, you can reply to this email and have your reply appear on GitHub as well. If your project does not have this feature enabled and wishes so, or if the feature is enabled but not working, please contact infrastructure at infrastruct...@apache.org or file a JIRA ticket with INFRA. ---
[jira] [Commented] (KAFKA-2389) CommitType seems not necessary in commit().
[ https://issues.apache.org/jira/browse/KAFKA-2389?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14694519#comment-14694519 ] ASF GitHub Bot commented on KAFKA-2389: --- GitHub user becketqin opened a pull request: https://github.com/apache/kafka/pull/134 KAFKA-2389: remove commit type from new consumer. A shot to remove commit type from new consumer. The coordinator constructor takes a default offset commit callback mainly for testing purpose. You can merge this pull request into a Git repository by running: $ git pull https://github.com/becketqin/kafka KAFKA-2389 Alternatively you can review and apply these changes as the patch at: https://github.com/apache/kafka/pull/134.patch To close this pull request, make a commit to your master/trunk branch with (at least) the following in the commit message: This closes #134 commit 5f332efa6c690fad730278283d8c419c6a223a8e Author: Jiangjie Qin becket@gmail.com Date: 2015-08-11T19:41:15Z KAFKA-2389: Remove commit type from commit() CommitType seems not necessary in commit(). --- Key: KAFKA-2389 URL: https://issues.apache.org/jira/browse/KAFKA-2389 Project: Kafka Issue Type: Sub-task Reporter: Jiangjie Qin Assignee: Jiangjie Qin The CommitType does not seem to be necessary in for commit(), it can be inferred from whether user passed in a callback or not. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
Re: Review Request 33049: Patch for KAFKA-2084
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/33049/#review95216 --- core/src/main/scala/kafka/server/ClientQuotaManager.scala (line 65) https://reviews.apache.org/r/33049/#comment150058 We can edit on check-in: should be `ClientQuotaManagerConfig` - Joel Koshy On Aug. 12, 2015, 7:09 p.m., Aditya Auradkar wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/33049/ --- (Updated Aug. 12, 2015, 7:09 p.m.) Review request for kafka, Joel Koshy and Jun Rao. Bugs: KAFKA-2084 https://issues.apache.org/jira/browse/KAFKA-2084 Repository: kafka Description --- Updated patch for quotas. This patch does the following: 1. Add per-client metrics for both producer and consumers 2. Add configuration for quotas 3. Compute delay times in the metrics package and return the delay times in QuotaViolationException 4. Add a DelayQueue in KafkaApi's that can be used to throttle any type of request. Implemented request throttling for produce and fetch requests. 5. Added unit and integration test cases for both producer and consumer 6. This doesn't include a system test. There is a separate ticket for that 7. Fixed KAFKA-2191 - (Included fix from : https://reviews.apache.org/r/34418/ ) Addressed comments from Joel and Jun Diffs - clients/src/main/java/org/apache/kafka/common/metrics/Quota.java d82bb0c055e631425bc1ebbc7d387baac76aeeaa clients/src/main/java/org/apache/kafka/common/metrics/QuotaViolationException.java a451e5385c9eca76b38b425e8ac856b2715fcffe clients/src/main/java/org/apache/kafka/common/metrics/Sensor.java ca823fd4639523018311b814fde69b6177e73b97 clients/src/main/java/org/apache/kafka/common/metrics/stats/Rate.java 98429da34418f7f1deba1b5e44e2e6025212edb3 clients/src/test/java/org/apache/kafka/common/metrics/MetricsTest.java 544e120594de78c43581a980b1e4087b4fb98ccb clients/src/test/java/org/apache/kafka/common/utils/MockTime.java core/src/main/scala/kafka/server/ClientQuotaManager.scala PRE-CREATION core/src/main/scala/kafka/server/KafkaApis.scala 7ea509c2c41acc00430c74e025e069a833aac4e7 core/src/main/scala/kafka/server/KafkaConfig.scala dbe170f87331f43e2dc30165080d2cb7dfe5fdbf core/src/main/scala/kafka/server/KafkaServer.scala 84d4730ac634f9a5bf12a656e422fea03ad72da8 core/src/main/scala/kafka/server/ReplicaManager.scala 795220e7f63d163be90738b4c1a39687b44c1395 core/src/main/scala/kafka/server/ThrottledResponse.scala PRE-CREATION core/src/main/scala/kafka/utils/ShutdownableThread.scala fc226c863095b7761290292cd8755cd7ad0f155c core/src/test/scala/integration/kafka/api/QuotasTest.scala PRE-CREATION core/src/test/scala/unit/kafka/server/ClientQuotaManagerTest.scala PRE-CREATION core/src/test/scala/unit/kafka/server/KafkaConfigTest.scala f32d206d3f52f3f9f4d649c213edd7058f4b6150 core/src/test/scala/unit/kafka/server/ThrottledResponseExpirationTest.scala PRE-CREATION Diff: https://reviews.apache.org/r/33049/diff/ Testing --- Thanks, Aditya Auradkar
[jira] [Commented] (KAFKA-2336) Changing offsets.topic.num.partitions after the offset topic is created breaks consumer group partition assignment
[ https://issues.apache.org/jira/browse/KAFKA-2336?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14694558#comment-14694558 ] Joel Koshy commented on KAFKA-2336: --- Sorry I should not have assigned it to myself before going on an extended leave of absence. Thanks [~granthenke] [~gwenshap] [~becket_qin] Changing offsets.topic.num.partitions after the offset topic is created breaks consumer group partition assignment --- Key: KAFKA-2336 URL: https://issues.apache.org/jira/browse/KAFKA-2336 Project: Kafka Issue Type: Bug Components: core Affects Versions: 0.8.2.1 Reporter: Grant Henke Assignee: Grant Henke Fix For: 0.8.3 Attachments: KAFKA-2336.patch, KAFKA-2336.patch, KAFKA-2336_2015-07-16_13:04:02.patch, KAFKA-2336_2015-08-11_10:37:41.patch Currently adjusting offsets.topic.num.partitions after the offset topic is created is not supported. Meaning that the number of partitions will not change once the topic has been created. However, changing the value in the configuration should not cause issues and instead simply be ignored. Currently this is not the case. When the value of offsets.topic.num.partitions is changed after the offset topic is created the consumer group partition assignment completely changes even though the number of partitions does not change. This is because _kafka.server.OffsetManager.partitionFor(group: String)_ uses the configured value and not the value of the actual topic. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Commented] (KAFKA-2425) Migrate website from SVN to Git
[ https://issues.apache.org/jira/browse/KAFKA-2425?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14694494#comment-14694494 ] Gwen Shapira commented on KAFKA-2425: - I guess I'm not imagining how it works correctly. Can you point me at a project that already does this, so I can see how it works? Migrate website from SVN to Git Key: KAFKA-2425 URL: https://issues.apache.org/jira/browse/KAFKA-2425 Project: Kafka Issue Type: Improvement Reporter: Ismael Juma The preference is to share the same Git repo for the code and website as per discussion in the mailing list: http://search-hadoop.com/m/uyzND1Dux842dm7vg2 Useful reference: https://blogs.apache.org/infra/entry/git_based_websites_available -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[GitHub] kafka pull request: KAFKA-Junit3 Misusage
Github user asfgit closed the pull request at: https://github.com/apache/kafka/pull/135 --- If your project is set up for it, you can reply to this email and have your reply appear on GitHub as well. If your project does not have this feature enabled and wishes so, or if the feature is enabled but not working, please contact infrastructure at infrastruct...@apache.org or file a JIRA ticket with INFRA. ---
Re: Review Request 33049: Patch for KAFKA-2084
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/33049/ --- (Updated Aug. 13, 2015, 4:24 a.m.) Review request for kafka, Joel Koshy and Jun Rao. Bugs: KAFKA-2084 https://issues.apache.org/jira/browse/KAFKA-2084 Repository: kafka Description (updated) --- Signed-off-by: Aditya Auradkar aaurad...@linkedin.com Addressing Joel's comments Minor imports changes Added testcase to verify that replication traffic is not throttled Tmp commit Fixing test failure Minor Addressing Joel's comments Addressing comments Addressing comments Addressing Juns comments Minor checkstyle changes fixed test case Addressing Juns comments Addressing Juns comments Addressing Juns comments Minor change Minor change Moving MockTime back to test Diffs (updated) - build.gradle 1b67e628c2fca897177c12b6afad9a8700fffd1f clients/src/main/java/org/apache/kafka/common/metrics/Quota.java d82bb0c055e631425bc1ebbc7d387baac76aeeaa clients/src/main/java/org/apache/kafka/common/metrics/QuotaViolationException.java a451e5385c9eca76b38b425e8ac856b2715fcffe clients/src/main/java/org/apache/kafka/common/metrics/Sensor.java ca823fd4639523018311b814fde69b6177e73b97 clients/src/main/java/org/apache/kafka/common/metrics/stats/Rate.java 98429da34418f7f1deba1b5e44e2e6025212edb3 clients/src/test/java/org/apache/kafka/common/metrics/MetricsTest.java 544e120594de78c43581a980b1e4087b4fb98ccb core/src/main/scala/kafka/server/ClientQuotaManager.scala PRE-CREATION core/src/main/scala/kafka/server/KafkaApis.scala 7ea509c2c41acc00430c74e025e069a833aac4e7 core/src/main/scala/kafka/server/KafkaConfig.scala dbe170f87331f43e2dc30165080d2cb7dfe5fdbf core/src/main/scala/kafka/server/KafkaServer.scala 84d4730ac634f9a5bf12a656e422fea03ad72da8 core/src/main/scala/kafka/server/ReplicaManager.scala 795220e7f63d163be90738b4c1a39687b44c1395 core/src/main/scala/kafka/server/ThrottledResponse.scala PRE-CREATION core/src/main/scala/kafka/utils/ShutdownableThread.scala fc226c863095b7761290292cd8755cd7ad0f155c core/src/test/scala/integration/kafka/api/QuotasTest.scala PRE-CREATION core/src/test/scala/unit/kafka/server/ClientQuotaManagerTest.scala PRE-CREATION core/src/test/scala/unit/kafka/server/KafkaConfigTest.scala f32d206d3f52f3f9f4d649c213edd7058f4b6150 core/src/test/scala/unit/kafka/server/ThrottledResponseExpirationTest.scala PRE-CREATION Diff: https://reviews.apache.org/r/33049/diff/ Testing --- Thanks, Aditya Auradkar
Re: NEW: API Stability annotations!
Good annotations. I can see a few future usages :) Jiangjie (Becket) Qin On Wed, Aug 12, 2015 at 3:05 PM, Gwen Shapira g...@confluent.io wrote: Hi Team Kafka, Ewen just added stability annotations to Apache Kafka (KAFKA-2429). In the same PR, we marked the new Consumer API as unstable since we are still actively iterating on them. The goal is to mark them as evolving before the next release and after one release to validate them, we will mark them as stable. When adding new public APIs, we encourage you to think of their stage of development and annotate correctly. The usage is: - unstable - can change at any time - evolving - can break compatibility at minor releases (i.e. 0.9 may be incompatible with 0.8) - stable - will only break compatibility at major releases (1.0, 2.0, etc) Enjoy! Gwen
Re: NEW: API Stability annotations!
Hi Gwen, Nice to see this. There is no deprecation cycle for breakages? Ismael On Wed, Aug 12, 2015 at 11:05 PM, Gwen Shapira g...@confluent.io wrote: Hi Team Kafka, Ewen just added stability annotations to Apache Kafka (KAFKA-2429). In the same PR, we marked the new Consumer API as unstable since we are still actively iterating on them. The goal is to mark them as evolving before the next release and after one release to validate them, we will mark them as stable. When adding new public APIs, we encourage you to think of their stage of development and annotate correctly. The usage is: - unstable - can change at any time - evolving - can break compatibility at minor releases (i.e. 0.9 may be incompatible with 0.8) - stable - will only break compatibility at major releases (1.0, 2.0, etc) Enjoy! Gwen
Re: Review Request 36858: Patch for KAFKA-2120
On Aug. 11, 2015, 8:49 p.m., Jason Gustafson wrote: clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerConfig.java, line 302 https://reviews.apache.org/r/36858/diff/4/?file=1037078#file1037078line302 Can we make this value greater than sessionTimeoutMs (which is 30s). Even if we don't address the issue of sanity between the different timeouts in this patch, it would be nice to have compatible defaults to keep the consumer from breaking out of the box. Mayuresh Gharat wrote: Hi Jason, I will upload a new patch with the sanity test. Mayuresh Gharat wrote: Hi Jason, I am creating a new jira ticket for the sanity test. This is because the requestTimeout in KafkaConsumer is long but it has to be int. This is because it is used as timeout in produceRequest that is used as replicationTimeout on the server side. The change might occur at multiple places and is completely orthogonal to this KIP. Here is the jira ticket : https://issues.apache.org/jira/browse/KAFKA-2428 Cool, thanks! - Jason --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/36858/#review94999 --- On Aug. 12, 2015, 5:59 p.m., Mayuresh Gharat wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/36858/ --- (Updated Aug. 12, 2015, 5:59 p.m.) Review request for kafka. Bugs: KAFKA-2120 https://issues.apache.org/jira/browse/KAFKA-2120 Repository: kafka Description --- Solved compile error Addressed Jason's comments for Kip-19 Addressed Jun's comments Addressed Jason's comments about the default values for requestTimeout Diffs - clients/src/main/java/org/apache/kafka/clients/ClientRequest.java dc8f0f115bcda893c95d17c0a57be8d14518d034 clients/src/main/java/org/apache/kafka/clients/CommonClientConfigs.java 2c421f42ed3fc5d61cf9c87a7eaa7bb23e26f63b clients/src/main/java/org/apache/kafka/clients/InFlightRequests.java 15d00d4e484bb5d51a9ae6857ed6e024a2cc1820 clients/src/main/java/org/apache/kafka/clients/KafkaClient.java 7ab2503794ff3aab39df881bd9fbae6547827d3b clients/src/main/java/org/apache/kafka/clients/NetworkClient.java 0e51d7bd461d253f4396a5b6ca7cd391658807fa clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerConfig.java d35b421a515074d964c7fccb73d260b847ea5f00 clients/src/main/java/org/apache/kafka/clients/consumer/KafkaConsumer.java ed99e9bdf7c4ea7a6d4555d4488cf8ed0b80641b clients/src/main/java/org/apache/kafka/clients/consumer/internals/ConsumerNetworkClient.java 9517d9d0cd480d5ba1d12f1fde7963e60528d2f8 clients/src/main/java/org/apache/kafka/clients/producer/KafkaProducer.java 03b8dd23df63a8d8a117f02eabcce4a2d48c44f7 clients/src/main/java/org/apache/kafka/clients/producer/ProducerConfig.java aa264202f2724907924985a5ecbe74afc4c6c04b clients/src/main/java/org/apache/kafka/clients/producer/internals/BufferPool.java 4cb1e50d6c4ed55241aeaef1d3af09def5274103 clients/src/main/java/org/apache/kafka/clients/producer/internals/RecordAccumulator.java a152bd7697dca55609a9ec4cfe0a82c10595fbc3 clients/src/main/java/org/apache/kafka/clients/producer/internals/RecordBatch.java 06182db1c3a5da85648199b4c0c98b80ea7c6c0c clients/src/main/java/org/apache/kafka/clients/producer/internals/Sender.java 0baf16e55046a2f49f6431e01d52c323c95eddf0 clients/src/main/java/org/apache/kafka/common/network/Selector.java ce20111ac434eb8c74585e9c63757bb9d60a832f clients/src/test/java/org/apache/kafka/clients/MockClient.java 9133d85342b11ba2c9888d4d2804d181831e7a8e clients/src/test/java/org/apache/kafka/clients/NetworkClientTest.java 43238ceaad0322e39802b615bb805b895336a009 clients/src/test/java/org/apache/kafka/clients/producer/internals/BufferPoolTest.java 2c693824fa53db1e38766b8c66a0ef42ef9d0f3a clients/src/test/java/org/apache/kafka/clients/producer/internals/RecordAccumulatorTest.java 5b2e4ffaeab7127648db608c179703b27b577414 clients/src/test/java/org/apache/kafka/clients/producer/internals/SenderTest.java 8b1805d3d2bcb9fe2bacb37d870c3236aa9532c4 clients/src/test/java/org/apache/kafka/common/network/SelectorTest.java 158f9829ff64a969008f699e40c51e918287859e core/src/main/scala/kafka/tools/ProducerPerformance.scala 0335cc64013ffe2cdf1c4879e86e11ec8c526712 core/src/test/scala/integration/kafka/api/ProducerFailureHandlingTest.scala ee94011894b46864614b97bbd2a98375a7d3f20b core/src/test/scala/unit/kafka/utils/TestUtils.scala eb169d8b33c27d598cc24e5a2e5f78b789fa38d3 Diff: https://reviews.apache.org/r/36858/diff/ Testing --- Thanks,
[jira] [Assigned] (KAFKA-2244) Document Kafka metrics configuration properties
[ https://issues.apache.org/jira/browse/KAFKA-2244?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Grant Henke reassigned KAFKA-2244: -- Assignee: Grant Henke Document Kafka metrics configuration properties --- Key: KAFKA-2244 URL: https://issues.apache.org/jira/browse/KAFKA-2244 Project: Kafka Issue Type: Task Components: config, website Affects Versions: 0.8.2.1 Reporter: Stevo Slavic Assignee: Grant Henke Labels: newbie Please have two configuration properties used in kafka.metrics.KafkaMetricsConfig, namely kafka.metrics.reporters and kafka.metrics.polling.interval.secs, documented on http://kafka.apache.org/documentation.html#configuration -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Updated] (KAFKA-2084) byte rate metrics per client ID (producer and consumer)
[ https://issues.apache.org/jira/browse/KAFKA-2084?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Aditya A Auradkar updated KAFKA-2084: - Attachment: KAFKA-2084_2015-08-12_21:24:07.patch byte rate metrics per client ID (producer and consumer) --- Key: KAFKA-2084 URL: https://issues.apache.org/jira/browse/KAFKA-2084 Project: Kafka Issue Type: Sub-task Reporter: Aditya Auradkar Assignee: Aditya Auradkar Labels: quotas Attachments: KAFKA-2084.patch, KAFKA-2084_2015-04-09_18:10:56.patch, KAFKA-2084_2015-04-10_17:24:34.patch, KAFKA-2084_2015-04-21_12:21:18.patch, KAFKA-2084_2015-04-21_12:28:05.patch, KAFKA-2084_2015-05-05_15:27:35.patch, KAFKA-2084_2015-05-05_17:52:02.patch, KAFKA-2084_2015-05-11_16:16:01.patch, KAFKA-2084_2015-05-26_11:50:50.patch, KAFKA-2084_2015-06-02_17:02:00.patch, KAFKA-2084_2015-06-02_17:09:28.patch, KAFKA-2084_2015-06-02_17:10:52.patch, KAFKA-2084_2015-06-04_16:31:22.patch, KAFKA-2084_2015-06-12_10:39:35.patch, KAFKA-2084_2015-06-29_17:53:44.patch, KAFKA-2084_2015-08-04_18:50:51.patch, KAFKA-2084_2015-08-04_19:07:46.patch, KAFKA-2084_2015-08-07_11:27:51.patch, KAFKA-2084_2015-08-10_13:48:50.patch, KAFKA-2084_2015-08-10_21:57:48.patch, KAFKA-2084_2015-08-12_12:02:33.patch, KAFKA-2084_2015-08-12_12:04:51.patch, KAFKA-2084_2015-08-12_12:08:17.patch, KAFKA-2084_2015-08-12_21:24:07.patch We need to be able to track the bytes-in/bytes-out rate on a per-client ID basis. This is necessary for quotas. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Commented] (KAFKA-2084) byte rate metrics per client ID (producer and consumer)
[ https://issues.apache.org/jira/browse/KAFKA-2084?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14694663#comment-14694663 ] Aditya A Auradkar commented on KAFKA-2084: -- Updated reviewboard https://reviews.apache.org/r/33049/diff/ against branch origin/trunk byte rate metrics per client ID (producer and consumer) --- Key: KAFKA-2084 URL: https://issues.apache.org/jira/browse/KAFKA-2084 Project: Kafka Issue Type: Sub-task Reporter: Aditya Auradkar Assignee: Aditya Auradkar Labels: quotas Attachments: KAFKA-2084.patch, KAFKA-2084_2015-04-09_18:10:56.patch, KAFKA-2084_2015-04-10_17:24:34.patch, KAFKA-2084_2015-04-21_12:21:18.patch, KAFKA-2084_2015-04-21_12:28:05.patch, KAFKA-2084_2015-05-05_15:27:35.patch, KAFKA-2084_2015-05-05_17:52:02.patch, KAFKA-2084_2015-05-11_16:16:01.patch, KAFKA-2084_2015-05-26_11:50:50.patch, KAFKA-2084_2015-06-02_17:02:00.patch, KAFKA-2084_2015-06-02_17:09:28.patch, KAFKA-2084_2015-06-02_17:10:52.patch, KAFKA-2084_2015-06-04_16:31:22.patch, KAFKA-2084_2015-06-12_10:39:35.patch, KAFKA-2084_2015-06-29_17:53:44.patch, KAFKA-2084_2015-08-04_18:50:51.patch, KAFKA-2084_2015-08-04_19:07:46.patch, KAFKA-2084_2015-08-07_11:27:51.patch, KAFKA-2084_2015-08-10_13:48:50.patch, KAFKA-2084_2015-08-10_21:57:48.patch, KAFKA-2084_2015-08-12_12:02:33.patch, KAFKA-2084_2015-08-12_12:04:51.patch, KAFKA-2084_2015-08-12_12:08:17.patch, KAFKA-2084_2015-08-12_21:24:07.patch We need to be able to track the bytes-in/bytes-out rate on a per-client ID basis. This is necessary for quotas. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[GitHub] kafka pull request: KAFKA-Junit3 Misusage
GitHub user ewencp opened a pull request: https://github.com/apache/kafka/pull/135 KAFKA-Junit3 Misusage You can merge this pull request into a Git repository by running: $ git pull https://github.com/ewencp/kafka kafka-1782-junit3-misusage Alternatively you can review and apply these changes as the patch at: https://github.com/apache/kafka/pull/135.patch To close this pull request, make a commit to your master/trunk branch with (at least) the following in the commit message: This closes #135 commit 0ae62589b81a9c0dc6be01b01092777923c5dae5 Author: Ewen Cheslack-Postava m...@ewencp.org Date: 2015-08-13T01:36:03Z KAFKA-1782: Junit3 Misusage --- If your project is set up for it, you can reply to this email and have your reply appear on GitHub as well. If your project does not have this feature enabled and wishes so, or if the feature is enabled but not working, please contact infrastructure at infrastruct...@apache.org or file a JIRA ticket with INFRA. ---
Re: Review Request 33049: Patch for KAFKA-2084
On June 17, 2015, 4:40 p.m., Joel Koshy wrote: core/src/main/scala/kafka/server/ClientQuotaMetrics.scala, line 1 https://reviews.apache.org/r/33049/diff/15/?file=983845#file983845line1 Why was MockTime moved from test to main? Aditya Auradkar wrote: Because I need to depend on MockTime from clients in core for unit tests written using the new metrics package. I don't think core depends on test from clients I think you can avoid this by spec `testCompile` depend on clients-test. I think that would be better - would prefer not allowing `MockTime` from leaking into the non-test packages. - Joel --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/33049/#review88180 --- On Aug. 12, 2015, 7:09 p.m., Aditya Auradkar wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/33049/ --- (Updated Aug. 12, 2015, 7:09 p.m.) Review request for kafka, Joel Koshy and Jun Rao. Bugs: KAFKA-2084 https://issues.apache.org/jira/browse/KAFKA-2084 Repository: kafka Description --- Updated patch for quotas. This patch does the following: 1. Add per-client metrics for both producer and consumers 2. Add configuration for quotas 3. Compute delay times in the metrics package and return the delay times in QuotaViolationException 4. Add a DelayQueue in KafkaApi's that can be used to throttle any type of request. Implemented request throttling for produce and fetch requests. 5. Added unit and integration test cases for both producer and consumer 6. This doesn't include a system test. There is a separate ticket for that 7. Fixed KAFKA-2191 - (Included fix from : https://reviews.apache.org/r/34418/ ) Addressed comments from Joel and Jun Diffs - clients/src/main/java/org/apache/kafka/common/metrics/Quota.java d82bb0c055e631425bc1ebbc7d387baac76aeeaa clients/src/main/java/org/apache/kafka/common/metrics/QuotaViolationException.java a451e5385c9eca76b38b425e8ac856b2715fcffe clients/src/main/java/org/apache/kafka/common/metrics/Sensor.java ca823fd4639523018311b814fde69b6177e73b97 clients/src/main/java/org/apache/kafka/common/metrics/stats/Rate.java 98429da34418f7f1deba1b5e44e2e6025212edb3 clients/src/test/java/org/apache/kafka/common/metrics/MetricsTest.java 544e120594de78c43581a980b1e4087b4fb98ccb clients/src/test/java/org/apache/kafka/common/utils/MockTime.java core/src/main/scala/kafka/server/ClientQuotaManager.scala PRE-CREATION core/src/main/scala/kafka/server/KafkaApis.scala 7ea509c2c41acc00430c74e025e069a833aac4e7 core/src/main/scala/kafka/server/KafkaConfig.scala dbe170f87331f43e2dc30165080d2cb7dfe5fdbf core/src/main/scala/kafka/server/KafkaServer.scala 84d4730ac634f9a5bf12a656e422fea03ad72da8 core/src/main/scala/kafka/server/ReplicaManager.scala 795220e7f63d163be90738b4c1a39687b44c1395 core/src/main/scala/kafka/server/ThrottledResponse.scala PRE-CREATION core/src/main/scala/kafka/utils/ShutdownableThread.scala fc226c863095b7761290292cd8755cd7ad0f155c core/src/test/scala/integration/kafka/api/QuotasTest.scala PRE-CREATION core/src/test/scala/unit/kafka/server/ClientQuotaManagerTest.scala PRE-CREATION core/src/test/scala/unit/kafka/server/KafkaConfigTest.scala f32d206d3f52f3f9f4d649c213edd7058f4b6150 core/src/test/scala/unit/kafka/server/ThrottledResponseExpirationTest.scala PRE-CREATION Diff: https://reviews.apache.org/r/33049/diff/ Testing --- Thanks, Aditya Auradkar
[jira] [Commented] (KAFKA-1782) Junit3 Misusage
[ https://issues.apache.org/jira/browse/KAFKA-1782?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14694602#comment-14694602 ] Guozhang Wang commented on KAFKA-1782: -- [~apakulov] Thanks for the patch, [~ewencp] helped fixing another issue and it has not been committed to trunk. Junit3 Misusage --- Key: KAFKA-1782 URL: https://issues.apache.org/jira/browse/KAFKA-1782 Project: Kafka Issue Type: Bug Reporter: Guozhang Wang Assignee: Alexander Pakulov Labels: newbie Fix For: 0.8.3 Attachments: KAFKA-1782.patch, KAFKA-1782.patch, KAFKA-1782_2015-06-18_11:52:49.patch, KAFKA-1782_2015-07-15_16:57:44.patch, KAFKA-1782_2015-07-16_11:50:05.patch, KAFKA-1782_2015-07-16_11:56:11.patch This is found while I was working on KAFKA-1580: in many of our cases where we explicitly extend from junit3suite (e.g. ProducerFailureHandlingTest), we are actually misusing a bunch of features that only exist in Junit4, such as (expected=classOf). For example, the following code {code} import org.scalatest.junit.JUnit3Suite import org.junit.Test import java.io.IOException class MiscTest extends JUnit3Suite { @Test (expected = classOf[IOException]) def testSendOffset() { } } {code} will actually pass even though IOException was not thrown since this annotation is not supported in Junit3. Whereas {code} import org.junit._ import java.io.IOException class MiscTest extends JUnit3Suite { @Test (expected = classOf[IOException]) def testSendOffset() { } } {code} or {code} import org.scalatest.junit.JUnitSuite import org.junit._ import java.io.IOException class MiscTest extends JUnit3Suite { @Test (expected = classOf[IOException]) def testSendOffset() { } } {code} or {code} import org.junit._ import java.io.IOException class MiscTest { @Test (expected = classOf[IOException]) def testSendOffset() { } } {code} will fail. I would propose to not rely on Junit annotations other than @Test itself but use scala unit test annotations instead, for example: {code} import org.junit._ import java.io.IOException class MiscTest { @Test def testSendOffset() { intercept[IOException] { //nothing } } } {code} will fail with a clearer stacktrace. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
Re: [DISCUSS] Client-side Assignment for New Consumer
On Tue, Aug 11, 2015 at 11:29 PM, Jiangjie Qin j...@linkedin.com.invalid wrote: Ewen, Thanks for the explanation. For (1), I am more concerned about the failure case instead of normal case. What if a consumer somehow was kick out of a group but is still consuming and committing offsets? Does that mean the new owner and old owner might potentially consuming from and committing offsets for the same partition? In the old consumer, this won't happen because the new consumer will not be able to start consumption unless the previous owner has released its ownership. Basically, without the ownership guarantee, I don't see how the communication among consumers themselves alone can solve the problem here. The generation ID check still applies to offset commits. If one of the consumers is kicked out and misbehaving, it can obviously still fetch and process messages, but offset commits will not work since it will not have the current generation ID. For (2) and (3), now I understand how metadata are used. But I still don't see why should we let the consumers to pass the topic information across instead of letting coordinator give the information. The single producer use case does not solve the ownership problem in abnormal case either, which seems to be a little bit vulnerable. One of the goals here was to generalize group membership so we can, for example, use it for balancing Copycat tasks across workers. There's no topic subscription info in that case. The metadata for copycat workers would instead need to somehow indicate the current set of tasks that need to be assigned to workers. By making the metadata completely opaque to the protocol, it becomes more generally useful since it focuses squarely on the group membership problem, allowing for that additional bit of metadata so you don't just get a list of members, but also get a little bit of info about each of them. A different option that we explored is to use a sort of mixed model -- still bake all the topic subscriptions directly into the protocol but also include metadata. That would allow us to maintain the existing coordinator-driven approach to handling the metadata and change events like the ones Onur pointed out. Then something like the Copycat workers would just not fill in any topic subscriptions and it would be handled as a degenerate case. Based on the way I explained that we can handle those types of events, I personally feel its cleaner and a nicer generalization to not include the subscriptions in the join group protocol, making it part of the metadata instead. For the single producer case, are you saying it doesn't solve ownership in the abnormal case because a producer that doesn't know it has been kicked out of the group yet can still produce data even though it shouldn't be able to anymore? I definitely agree that that is a risk -- this provides a way to get closer to a true single-writer, but there are definitely still failure modes that this does not address. -Ewen Thanks, Jiangjie (Becket) Qin On Tue, Aug 11, 2015 at 11:06 PM, Ewen Cheslack-Postava e...@confluent.io wrote: On Tue, Aug 11, 2015 at 10:15 PM, Jiangjie Qin j...@linkedin.com.invalid wrote: Hi Jason, Thanks for writing this up. It would be useful to generalize the group concept. I have a few questions below. 1. In old consumer actually the partition assignment are done by consumers themselves. We used zookeeper to guarantee that a partition will only be consumed by one consumer thread who successfully claimed its ownership. Does the new protocol plan to provide the same guarantee? Once you have all the metadata from all the consumers, assignment should just be a simple function mapping that MapConsumerId, Metadata to MapConsumerId, ListTopicPartition. If everyone is consistent in computing that, you don't need ZK involved at all. In practice, this shouldn't be that hard to ensure for most assignment strategies just by having decent unit testing on them. You just have to do things like ensure your assignment strategy sorts lists into a consistent order. You do give up the ability to use some techniques (e.g. any randomized algorithm if you can't distribute the seed w/ the metadata) and it's true that nothing validates the assignment, but if that assignment algorithm step is kept simple, small, and well tested, the risk is very minimal. 2. It looks that both JoinGroupRequest and JoinGroupResponse has the ProtocolMetadata.AssignmentStrategyMetadata, what would be the metadata be sent and returned by coordinator? How will the coordinator handle the metadata? The coordinator is basically just blindly broadcasting all of it to group members so they have a consistent view. So from the coordinators perspective, it sees something like: Consumer 1 - JoinGroupRequest with GroupProtocols = [ consumer Consumer1 opaque byte[]] Consumer 2 -
[jira] [Created] (KAFKA-2423) Introduce Scalastyle
Ismael Juma created KAFKA-2423: -- Summary: Introduce Scalastyle Key: KAFKA-2423 URL: https://issues.apache.org/jira/browse/KAFKA-2423 Project: Kafka Issue Type: Improvement Reporter: Ismael Juma Assignee: Ismael Juma This is similar to Checkstyle (which we already use), but for Scala: http://www.scalastyle.org/ -- This message was sent by Atlassian JIRA (v6.3.4#6332)
Re: Kafka Indentation
On Wed, Aug 12, 2015 at 1:29 AM, Gwen Shapira g...@confluent.io wrote: +1 on not breaking git blame -1 on rewriting Kafka in Java +1 on upping our Scala game (as Ismael pointed out) I filed a couple of JIRAs, and I'll look at introducing Scalastyle once some of the bigger patches/PRs are merged. https://issues.apache.org/jira/browse/KAFKA-2423 https://issues.apache.org/jira/browse/KAFKA-2424 Regarding the rewrite of Kafka in Java, I think it's a bit early to be voting on that given that Jay's suggestion was to do it once we can use Java 8 in the broker (over a year away). I'm with you Gwen, let's up our Scala game in the meantime. Once the time comes to make the decision, we can evaluate the pros and cons and have the vote. Ismael
[jira] [Commented] (KAFKA-2364) Improve documentation for contributing to docs
[ https://issues.apache.org/jira/browse/KAFKA-2364?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14693167#comment-14693167 ] Ismael Juma commented on KAFKA-2364: [~anshbansal], the consensus was that moving to Git was good and the website should live in the same repo as the kafka code. I filed KAFKA-2425 for it. Improve documentation for contributing to docs -- Key: KAFKA-2364 URL: https://issues.apache.org/jira/browse/KAFKA-2364 Project: Kafka Issue Type: Task Reporter: Aseem Bansal Priority: Minor Labels: doc While reading the documentation for kafka 8 I saw some improvements that can be made. But the docs for contributing are not very good at https://github.com/apache/kafka. It just gives me a URL for svn. But I am not sure what to do. Can the README.MD file be improved for contributing to docs? I have submitted patches to groovy and grails by sending PRs via github but looking at the comments on PRs submitted to kafak it seems PRs via github are not working for kafka. It would be good to make that work also. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
Re: [DISCUSS] Client-side Assignment for New Consumer
On Tue, Aug 11, 2015 at 10:15 PM, Jiangjie Qin j...@linkedin.com.invalid wrote: Hi Jason, Thanks for writing this up. It would be useful to generalize the group concept. I have a few questions below. 1. In old consumer actually the partition assignment are done by consumers themselves. We used zookeeper to guarantee that a partition will only be consumed by one consumer thread who successfully claimed its ownership. Does the new protocol plan to provide the same guarantee? Once you have all the metadata from all the consumers, assignment should just be a simple function mapping that MapConsumerId, Metadata to MapConsumerId, ListTopicPartition. If everyone is consistent in computing that, you don't need ZK involved at all. In practice, this shouldn't be that hard to ensure for most assignment strategies just by having decent unit testing on them. You just have to do things like ensure your assignment strategy sorts lists into a consistent order. You do give up the ability to use some techniques (e.g. any randomized algorithm if you can't distribute the seed w/ the metadata) and it's true that nothing validates the assignment, but if that assignment algorithm step is kept simple, small, and well tested, the risk is very minimal. 2. It looks that both JoinGroupRequest and JoinGroupResponse has the ProtocolMetadata.AssignmentStrategyMetadata, what would be the metadata be sent and returned by coordinator? How will the coordinator handle the metadata? The coordinator is basically just blindly broadcasting all of it to group members so they have a consistent view. So from the coordinators perspective, it sees something like: Consumer 1 - JoinGroupRequest with GroupProtocols = [ consumer Consumer1 opaque byte[]] Consumer 2 - JoinGroupRequest with GroupProtocols = [ consumer Consumer2 opaque byte[]] Then, in the responses would look like: Consumer 1 - JoinGroupResponse with GroupProtocol = consumer and GroupMembers = [ Consumer 1 Consumer1 opaque byte[], Consumer 2 Consumer2 opaque byte[]] Consumer 2 - JoinGroupResponse with GroupProtocol = consumer and GroupMembers = [ Consumer 1 Consumer1 opaque byte[], Consumer 2 Consumer2 opaque byte[]] So all the responses include all the metadata for every member in the group, and everyone can use that to consistently decide on assignment. The broker doesn't care and cannot even understand the metadata since the data format for it is dependent on the assignment strategy being used. As another example that is *not* a consumer, let's say you just want to have a single writer in the group which everyone will forward requests to. To accomplish this, you could use a very dumb assignment strategy: there is no metadata (empty byte[]) and all we care about is who is the first member in the group (e.g. when IDs are sorted lexicographically). That member is selected as the writer. In that case, we actually just care about the membership list, there's no additional info about each member that is required to determine who is the writer. 3. Do you mean that the number of partitions in JoinGroupResponse will be the max partition number of a topic among all the reported partition number by consumers? Is there any reason not just let Coordinator to return the number of partitions of a topic in its metadata cache? Nothing from the embedded protocol is touched by the broker. The broker just collects opaque bytes of metadata, does the selection of the strategy if multiple are supported by some consumers, and then returns that opaque metadata for all the members back to every member. In that way they all have a consistent view of the group. For regular consumers, that view of the group includes information about how many partitions each consumer currently thinks the topics it is subscribed to has. These could be inconsistent due to out of date metadata and it would be up to the assignment strategy on the *client* to resolve that. As you point out, in that case they could just take the max value that any consumer reported seeing and use that. The consumers that notice that their metadata had a smaller # of partitions should also trigger a metadata update when they see someone else observing a larger # of partitions. Thanks, Jiangjie (Becket) Qin On Tue, Aug 11, 2015 at 1:19 PM, Jason Gustafson ja...@confluent.io wrote: Hi Kafka Devs, One of the nagging issues in the current design of the new consumer has been the need to support a variety of assignment strategies. We've encountered this in particular in the design of copycat and the processing framework (KIP-28). From what I understand, Samza also has a number of use cases with custom assignment needs. The new consumer protocol supports new assignment strategies by hooking them into the broker. For many environments, this is a major pain and in some cases, a non-starter. It also challenges the validation that the coordinator can provide. For example, some assignment
Re: Kafka Indentation
On Wed, Aug 12, 2015 at 1:23 AM, Jason Gustafson ja...@confluent.io wrote: Can the java code be indented without affecting the results of git blame? If not, then I'd vote to leave it as it is. Nope. Ismael
[jira] [Created] (KAFKA-2424) Consider introducing lint-like tool for Scala
Ismael Juma created KAFKA-2424: -- Summary: Consider introducing lint-like tool for Scala Key: KAFKA-2424 URL: https://issues.apache.org/jira/browse/KAFKA-2424 Project: Kafka Issue Type: Improvement Reporter: Ismael Juma Typesafe is working on abide and the first release is expected next month: https://github.com/scala/scala-abide An alternative is scapegoat: https://github.com/sksamuel/scalac-scapegoat-plugin -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Updated] (KAFKA-2424) Consider introducing lint-like tool for Scala
[ https://issues.apache.org/jira/browse/KAFKA-2424?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Ismael Juma updated KAFKA-2424: --- Labels: newbie (was: ) Consider introducing lint-like tool for Scala - Key: KAFKA-2424 URL: https://issues.apache.org/jira/browse/KAFKA-2424 Project: Kafka Issue Type: Improvement Reporter: Ismael Juma Labels: newbie Typesafe is working on abide and the first release is expected next month: https://github.com/scala/scala-abide An alternative is scapegoat: https://github.com/sksamuel/scalac-scapegoat-plugin -- This message was sent by Atlassian JIRA (v6.3.4#6332)
Re: KAFKA-2364 migrate docs from SVN to git
Hi Gwen, I filed KAFKA-2425 as KAFKA-2364 is about improving the website documentation. Aseem Bansal seemed interested in helping us with the move so I pinged him in the issue. Best, Ismael On Wed, Aug 12, 2015 at 1:51 AM, Gwen Shapira g...@confluent.io wrote: Ah, there is already a JIRA in the title. Never mind :) On Tue, Aug 11, 2015 at 5:51 PM, Gwen Shapira g...@confluent.io wrote: The vote opened 5 days ago. I believe we can conclude with 3 binding +1, 3 non-binding +1 and no -1. Ismael, are you opening and JIRA and migrating? Or are we looking for a volunteer? On Tue, Aug 11, 2015 at 5:46 PM, Ashish Singh asi...@cloudera.com wrote: +1 on same repo. On Tue, Aug 11, 2015 at 12:21 PM, Edward Ribeiro edward.ribe...@gmail.com wrote: +1. As soon as possible, please. :) On Sat, Aug 8, 2015 at 4:05 PM, Neha Narkhede n...@confluent.io wrote: +1 on the same repo for code and website. It helps to keep both in sync. On Thu, Aug 6, 2015 at 1:52 PM, Grant Henke ghe...@cloudera.com wrote: +1 for the same repo. The closer docs can be to code the more accurate they are likely to be. The same way we encourage unit tests for a new feature/patch. Updating the docs can be the same. If we follow Sqoop's process for example, how would small fixes/adjustments/additions to the live documentation occur without a new release? On Thu, Aug 6, 2015 at 3:33 PM, Guozhang Wang wangg...@gmail.com wrote: I am +1 on same repo too. I think keeping one git history of code / doc change may actually be beneficial for this approach as well. Guozhang On Thu, Aug 6, 2015 at 9:16 AM, Gwen Shapira g...@confluent.io wrote: I prefer same repo for one-commit / lower-barrier benefits. Sqoop has the following process, which decouples documentation changes from website changes: 1. Code github repo contains a doc directory, with the documentation written and maintained in AsciiDoc. Only one version of the documentation, since it is source controlled with the code. (unlike current SVN where we have directories per version) 2. Build process compiles the AsciiDoc to HTML and PDF 3. When releasing, we post the documentation of the new release to the website Gwen On Thu, Aug 6, 2015 at 12:20 AM, Ismael Juma ism...@juma.me.uk wrote: Hi, For reference, here is the previous discussion on moving the website to Git: http://search-hadoop.com/m/uyzND11JliU1E8QU92 People were positive to the idea as Jay said. I would like to see a bit of a discussion around whether the website should be part of the same repo as the code or not. I'll get the ball rolling. Pros for same repo: * One commit can update the code and website, which means: ** Lower barrier for updating docs along with relevant code changes ** Easier to require that both are updated at the same time * More eyeballs on the website changes * Automatically branched with the relevant code Pros for separate repo: * Potentially simpler for website-only changes (smaller repo, less verification needed) * Website changes don't clutter the code Git history * No risk of website change affecting the code Your thoughts, please. Best, Ismael On Fri, Jul 31, 2015 at 6:15 PM, Aseem Bansal asmbans...@gmail.com wrote: Hi When discussing on KAFKA-2364 migrating docs from svn to git came up. That would make contributing to docs much easier. I have contributed to groovy/grails via github so I think having mirror on github could be useful. Also I think unless there is some good reason it should be a separate repo. No need to mix docs and code. I can try that out. Thoughts? -- -- Guozhang -- Grant Henke Software Engineer | Cloudera gr...@cloudera.com | twitter.com/gchenke | linkedin.com/in/granthenke -- Thanks, Neha -- Regards, Ashish
[jira] [Commented] (KAFKA-2165) ReplicaFetcherThread: data loss on unknown exception
[ https://issues.apache.org/jira/browse/KAFKA-2165?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14693242#comment-14693242 ] Artem Kirillov commented on KAFKA-2165: --- We experience the same issue on our production cluster built on Kafka 0.8.2.1. But why this was resolved as not a problem? I can't imagine situation when during normal work replica is getting ahead leader. It's possible only if leader loses some messages already replicated by replica. ReplicaFetcherThread: data loss on unknown exception Key: KAFKA-2165 URL: https://issues.apache.org/jira/browse/KAFKA-2165 Project: Kafka Issue Type: Bug Affects Versions: 0.8.2.1 Reporter: Alexey Ozeritskiy Attachments: KAFKA-2165.patch Sometimes in our cluster some replica gets out of the isr. Then broker redownloads the partition from the beginning. We got the following messages in logs: {code} # The leader: [2015-03-25 11:11:07,796] ERROR [Replica Manager on Broker 21]: Error when processing fetch request for partition [topic,11] offset 54369274 from follower with correlation id 2634499. Possible cause: Request for offset 54369274 but we only have log segments in the range 49322124 to 54369273. (kafka.server.ReplicaManager) {code} {code} # The follower: [2015-03-25 11:11:08,816] WARN [ReplicaFetcherThread-0-21], Replica 31 for partition [topic,11] reset its fetch offset from 49322124 to current leader 21's start offset 49322124 (kafka.server.ReplicaFetcherThread) [2015-03-25 11:11:08,816] ERROR [ReplicaFetcherThread-0-21], Current offset 54369274 for partition [topic,11] out of range; reset offset to 49322124 (kafka.server.ReplicaFetcherThread) {code} This occures because we update fetchOffset [here|https://github.com/apache/kafka/blob/0.8.2/core/src/main/scala/kafka/server/AbstractFetcherThread.scala#L124] and then try to process message. If any exception except OffsetOutOfRangeCode occures we get unsynchronized fetchOffset and replica.logEndOffset. On next fetch iteration we can get fetchOffsetreplica.logEndOffset==leaderEndOffset and OffsetOutOfRangeCode. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Created] (KAFKA-2427) Error writing to highwatermark file
Harish Seshadri created KAFKA-2427: -- Summary: Error writing to highwatermark file Key: KAFKA-2427 URL: https://issues.apache.org/jira/browse/KAFKA-2427 Project: Kafka Issue Type: Bug Components: replication Affects Versions: 0.8.2.1 Environment: Ubuntu 14.04 Reporter: Harish Seshadri Assignee: Neha Narkhede Priority: Critical Periodically one instance of the kafka broker crashes (process exits) with the following error. Note: The persistence of files makes use of NFS mount [2015-08-12 08:42:12,480] FATAL [Replica Manager on Broker 1]: Error writing to highwatermark file: (kafka.server.ReplicaManager) java.io.IOException: File rename from /nfs/data/kafka1-logs/replication-offset-checkpoint.tmp to /nfs/data/kafka1-logs/replication-offset-checkpoint failed. at kafka.server.OffsetCheckpoint.write(OffsetCheckpoint.scala:66) at kafka.server.ReplicaManager$$anonfun$checkpointHighWatermarks$2.apply(ReplicaManager.scala:596) at kafka.server.ReplicaManager$$anonfun$checkpointHighWatermarks$2.apply(ReplicaManager.scala:593) at scala.collection.TraversableLike$WithFilter$$anonfun$foreach$1.apply(TraversableLike.scala:772) at scala.collection.immutable.Map$Map1.foreach(Map.scala:109) at scala.collection.TraversableLike$WithFilter.foreach(TraversableLike.scala:771) at kafka.server.ReplicaManager.checkpointHighWatermarks(ReplicaManager.scala:593) at kafka.server.ReplicaManager$$anonfun$1.apply$mcV$sp(ReplicaManager.scala:99) at kafka.utils.KafkaScheduler$$anonfun$1.apply$mcV$sp(KafkaScheduler.scala:99) at kafka.utils.Utils$$anon$1.run(Utils.scala:54) at java.util.concurrent.Executors$RunnableAdapter.call(Executors.java:471) at java.util.concurrent.FutureTask.runAndReset(FutureTask.java:304) at java.util.concurrent.ScheduledThreadPoolExecutor$ScheduledFutureTask.access$301(ScheduledThreadPoolExecutor.java:178) at java.util.concurrent.ScheduledThreadPoolExecutor$ScheduledFutureTask.run(ScheduledThreadPoolExecutor.java:293) at java.util.concurrent.ThreadPoolExecutor.runWorker(ThreadPoolExecutor.java:1145) at java.util.concurrent.ThreadPoolExecutor$Worker.run(ThreadPoolExecutor.java:615) at java.lang.Thread.run(Thread.java:745) (END)packet_write_wait: Connection to 10.23.2.110: Broken pipe -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Commented] (KAFKA-2425) Migrate website from SVN to Git
[ https://issues.apache.org/jira/browse/KAFKA-2425?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14693330#comment-14693330 ] Manikumar Reddy commented on KAFKA-2425: You can find the docs here : http://svn.apache.org/repos/asf/kafka/site Things to consider/to do: # Raise an Apache INFRA ticket to finalize the migration process. INFRA prefers to new asf-site branch for site docs.. # checkout the site docs from svn and remove all .svn folders # open a github pull request and commit the site docs # Inform INFRA team to complete the migration. Migrate website from SVN to Git Key: KAFKA-2425 URL: https://issues.apache.org/jira/browse/KAFKA-2425 Project: Kafka Issue Type: Improvement Reporter: Ismael Juma The preference is to share the same Git repo for the code and website as per discussion in the mailing list: http://search-hadoop.com/m/uyzND1Dux842dm7vg2 Useful reference: https://blogs.apache.org/infra/entry/git_based_websites_available -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Commented] (KAFKA-2425) Migrate website from SVN to Git
[ https://issues.apache.org/jira/browse/KAFKA-2425?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14693182#comment-14693182 ] Ismael Juma commented on KAFKA-2425: [~anshbansal], are you interested in attempting this? Migrate website from SVN to Git Key: KAFKA-2425 URL: https://issues.apache.org/jira/browse/KAFKA-2425 Project: Kafka Issue Type: Improvement Reporter: Ismael Juma The preference is to share the same Git repo for the code and website as per discussion in the mailing list: http://search-hadoop.com/m/uyzND1Dux842dm7vg2 Useful reference: https://blogs.apache.org/infra/entry/git_based_websites_available -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Comment Edited] (KAFKA-2078) Getting Selector [WARN] Error in I/O with host java.io.EOFException
[ https://issues.apache.org/jira/browse/KAFKA-2078?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14693213#comment-14693213 ] PC edited comment on KAFKA-2078 at 8/12/15 9:30 AM: Here is an update on what appears to cause this problem: Pumping 4 messages, 1 from each of the 4 producers: 11:16:26.333 [kafka-producer-network-thread | producer-1] DEBUG producer - PumpSuccess topic[test] partition[0] offset[582177743] 11:16:26.333 [kafka-producer-network-thread | producer-3] DEBUG producer - PumpSuccess topic[test] partition[0] offset[582177741] 11:16:26.333 [kafka-producer-network-thread | producer-2] DEBUG producer - PumpSuccess topic[test] partition[0] offset[582177744] 11:16:26.333 [kafka-producer-network-thread | producer-4] DEBUG producer - PumpSuccess topic[test] partition[0] offset[582177742] 11:21:26.243 [kafka-producer-network-thread | producer-3] DEBUG o.apache.kafka.clients.NetworkClient - Trying to send metadata request to node 0 11:21:26.243 [kafka-producer-network-thread | producer-4] DEBUG o.apache.kafka.clients.NetworkClient - Trying to send metadata request to node 0 11:21:26.243 [kafka-producer-network-thread | producer-1] DEBUG o.apache.kafka.clients.NetworkClient - Trying to send metadata request to node 0 11:21:26.243 [kafka-producer-network-thread | producer-2] DEBUG o.apache.kafka.clients.NetworkClient - Trying to send metadata request to node 0 11:21:26.243 [kafka-producer-network-thread | producer-3] DEBUG o.apache.kafka.clients.NetworkClient - Sending metadata request ClientRequest(expectResponse=true, payload=null, request=RequestSend(header={api_key=3,api_version=0,correlation_id=2,client_id=producer-3}, body={topics=[test]})) to node 0 11:21:26.243 [kafka-producer-network-thread | producer-4] DEBUG o.apache.kafka.clients.NetworkClient - Sending metadata request ClientRequest(expectResponse=true, payload=null, request=RequestSend(header={api_key=3,api_version=0,correlation_id=2,client_id=producer-4}, body={topics=[test]})) to node 0 11:21:26.244 [kafka-producer-network-thread | producer-2] DEBUG o.apache.kafka.clients.NetworkClient - Sending metadata request ClientRequest(expectResponse=true, payload=null, request=RequestSend(header={api_key=3,api_version=0,correlation_id=2,client_id=producer-2}, body={topics=[test]})) to node 0 11:21:26.244 [kafka-producer-network-thread | producer-1] DEBUG o.apache.kafka.clients.NetworkClient - Sending metadata request ClientRequest(expectResponse=true, payload=null, request=RequestSend(header={api_key=3,api_version=0,correlation_id=2,client_id=producer-1}, body={topics=[test]})) to node 0 11:21:26.245 [kafka-producer-network-thread | producer-2] DEBUG o.a.k.c.producer.internals.Metadata - Updated cluster metadata version 3 to Cluster(nodes = [Node(0, localhost, 9092)], partitions = [Partition(topic = test, partition = 0, leader = 0, replicas = [0,], isr = [0,]]) 11:21:26.245 [kafka-producer-network-thread | producer-3] DEBUG o.a.k.c.producer.internals.Metadata - Updated cluster metadata version 3 to Cluster(nodes = [Node(0, localhost, 9092)], partitions = [Partition(topic = test, partition = 0, leader = 0, replicas = [0,], isr = [0,]]) 11:21:26.245 [kafka-producer-network-thread | producer-1] DEBUG o.a.k.c.producer.internals.Metadata - Updated cluster metadata version 3 to Cluster(nodes = [Node(0, localhost, 9092)], partitions = [Partition(topic = test, partition = 0, leader = 0, replicas = [0,], isr = [0,]]) 11:21:26.245 [kafka-producer-network-thread | producer-4] DEBUG o.a.k.c.producer.internals.Metadata - Updated cluster metadata version 3 to Cluster(nodes = [Node(0, localhost, 9092)], partitions = [Partition(topic = test, partition = 0, leader = 0, replicas = [0,], isr = [0,]]) 11:21:26.243 [kafka-producer-network-thread | producer-3] DEBUG o.apache.kafka.clients.NetworkClient - Trying to send metadata request to node 0 11:21:26.243 [kafka-producer-network-thread | producer-4] DEBUG o.apache.kafka.clients.NetworkClient - Trying to send metadata request to node 0 11:21:26.243 [kafka-producer-network-thread | producer-1] DEBUG o.apache.kafka.clients.NetworkClient - Trying to send metadata request to node 0 11:21:26.243 [kafka-producer-network-thread | producer-2] DEBUG o.apache.kafka.clients.NetworkClient - Trying to send metadata request to node 0 11:21:26.243 [kafka-producer-network-thread | producer-3] DEBUG o.apache.kafka.clients.NetworkClient - Sending metadata request ClientRequest(expectResponse=true, payload=null, request=RequestSend(header={api_key=3,api_version=0,correlation_id=2,client_id=producer-3}, body={topics=[test]})) to node 0 11:21:26.243 [kafka-producer-network-thread | producer-4] DEBUG o.apache.kafka.clients.NetworkClient - Sending metadata request ClientRequest(expectResponse=true, payload=null,
[jira] [Commented] (KAFKA-2078) Getting Selector [WARN] Error in I/O with host java.io.EOFException
[ https://issues.apache.org/jira/browse/KAFKA-2078?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14693213#comment-14693213 ] PC commented on KAFKA-2078: --- Here is an update on appears to cause this problem: Pumping 4 messages, 1 from each of the 4 producers: 11:16:26.333 [kafka-producer-network-thread | producer-1] DEBUG producer - PumpSuccess topic[test] partition[0] offset[582177743] 11:16:26.333 [kafka-producer-network-thread | producer-3] DEBUG producer - PumpSuccess topic[test] partition[0] offset[582177741] 11:16:26.333 [kafka-producer-network-thread | producer-2] DEBUG producer - PumpSuccess topic[test] partition[0] offset[582177744] 11:16:26.333 [kafka-producer-network-thread | producer-4] DEBUG producer - PumpSuccess topic[test] partition[0] offset[582177742] 11:21:26.243 [kafka-producer-network-thread | producer-3] DEBUG o.apache.kafka.clients.NetworkClient - Trying to send metadata request to node 0 11:21:26.243 [kafka-producer-network-thread | producer-4] DEBUG o.apache.kafka.clients.NetworkClient - Trying to send metadata request to node 0 11:21:26.243 [kafka-producer-network-thread | producer-1] DEBUG o.apache.kafka.clients.NetworkClient - Trying to send metadata request to node 0 11:21:26.243 [kafka-producer-network-thread | producer-2] DEBUG o.apache.kafka.clients.NetworkClient - Trying to send metadata request to node 0 11:21:26.243 [kafka-producer-network-thread | producer-3] DEBUG o.apache.kafka.clients.NetworkClient - Sending metadata request ClientRequest(expectResponse=true, payload=null, request=RequestSend(header={api_key=3,api_version=0,correlation_id=2,client_id=producer-3}, body={topics=[test]})) to node 0 11:21:26.243 [kafka-producer-network-thread | producer-4] DEBUG o.apache.kafka.clients.NetworkClient - Sending metadata request ClientRequest(expectResponse=true, payload=null, request=RequestSend(header={api_key=3,api_version=0,correlation_id=2,client_id=producer-4}, body={topics=[test]})) to node 0 11:21:26.244 [kafka-producer-network-thread | producer-2] DEBUG o.apache.kafka.clients.NetworkClient - Sending metadata request ClientRequest(expectResponse=true, payload=null, request=RequestSend(header={api_key=3,api_version=0,correlation_id=2,client_id=producer-2}, body={topics=[test]})) to node 0 11:21:26.244 [kafka-producer-network-thread | producer-1] DEBUG o.apache.kafka.clients.NetworkClient - Sending metadata request ClientRequest(expectResponse=true, payload=null, request=RequestSend(header={api_key=3,api_version=0,correlation_id=2,client_id=producer-1}, body={topics=[test]})) to node 0 11:21:26.245 [kafka-producer-network-thread | producer-2] DEBUG o.a.k.c.producer.internals.Metadata - Updated cluster metadata version 3 to Cluster(nodes = [Node(0, localhost, 9092)], partitions = [Partition(topic = test, partition = 0, leader = 0, replicas = [0,], isr = [0,]]) 11:21:26.245 [kafka-producer-network-thread | producer-3] DEBUG o.a.k.c.producer.internals.Metadata - Updated cluster metadata version 3 to Cluster(nodes = [Node(0, localhost, 9092)], partitions = [Partition(topic = test, partition = 0, leader = 0, replicas = [0,], isr = [0,]]) 11:21:26.245 [kafka-producer-network-thread | producer-1] DEBUG o.a.k.c.producer.internals.Metadata - Updated cluster metadata version 3 to Cluster(nodes = [Node(0, localhost, 9092)], partitions = [Partition(topic = test, partition = 0, leader = 0, replicas = [0,], isr = [0,]]) 11:21:26.245 [kafka-producer-network-thread | producer-4] DEBUG o.a.k.c.producer.internals.Metadata - Updated cluster metadata version 3 to Cluster(nodes = [Node(0, localhost, 9092)], partitions = [Partition(topic = test, partition = 0, leader = 0, replicas = [0,], isr = [0,]]) 11:21:26.243 [kafka-producer-network-thread | producer-3] DEBUG o.apache.kafka.clients.NetworkClient - Trying to send metadata request to node 0 11:21:26.243 [kafka-producer-network-thread | producer-4] DEBUG o.apache.kafka.clients.NetworkClient - Trying to send metadata request to node 0 11:21:26.243 [kafka-producer-network-thread | producer-1] DEBUG o.apache.kafka.clients.NetworkClient - Trying to send metadata request to node 0 11:21:26.243 [kafka-producer-network-thread | producer-2] DEBUG o.apache.kafka.clients.NetworkClient - Trying to send metadata request to node 0 11:21:26.243 [kafka-producer-network-thread | producer-3] DEBUG o.apache.kafka.clients.NetworkClient - Sending metadata request ClientRequest(expectResponse=true, payload=null, request=RequestSend(header={api_key=3,api_version=0,correlation_id=2,client_id=producer-3}, body={topics=[test]})) to node 0 11:21:26.243 [kafka-producer-network-thread | producer-4] DEBUG o.apache.kafka.clients.NetworkClient - Sending metadata request ClientRequest(expectResponse=true, payload=null, request=RequestSend(header={api_key=3,api_version=0,correlation_id=2,client_id=producer-4}, body={topics=[test]})) to node 0
Re: KAFKA-2364 migrate docs from SVN to git
FYI, I created a tiny trivial patch to address a typo in the web site (KAFKA-2418), so maybe you can review it and eventually commit before moving to github. ;) Cheers, Eddie Em 12/08/2015 06:01, Ismael Juma ism...@juma.me.uk escreveu: Hi Gwen, I filed KAFKA-2425 as KAFKA-2364 is about improving the website documentation. Aseem Bansal seemed interested in helping us with the move so I pinged him in the issue. Best, Ismael On Wed, Aug 12, 2015 at 1:51 AM, Gwen Shapira g...@confluent.io wrote: Ah, there is already a JIRA in the title. Never mind :) On Tue, Aug 11, 2015 at 5:51 PM, Gwen Shapira g...@confluent.io wrote: The vote opened 5 days ago. I believe we can conclude with 3 binding +1, 3 non-binding +1 and no -1. Ismael, are you opening and JIRA and migrating? Or are we looking for a volunteer? On Tue, Aug 11, 2015 at 5:46 PM, Ashish Singh asi...@cloudera.com wrote: +1 on same repo. On Tue, Aug 11, 2015 at 12:21 PM, Edward Ribeiro edward.ribe...@gmail.com wrote: +1. As soon as possible, please. :) On Sat, Aug 8, 2015 at 4:05 PM, Neha Narkhede n...@confluent.io wrote: +1 on the same repo for code and website. It helps to keep both in sync. On Thu, Aug 6, 2015 at 1:52 PM, Grant Henke ghe...@cloudera.com wrote: +1 for the same repo. The closer docs can be to code the more accurate they are likely to be. The same way we encourage unit tests for a new feature/patch. Updating the docs can be the same. If we follow Sqoop's process for example, how would small fixes/adjustments/additions to the live documentation occur without a new release? On Thu, Aug 6, 2015 at 3:33 PM, Guozhang Wang wangg...@gmail.com wrote: I am +1 on same repo too. I think keeping one git history of code / doc change may actually be beneficial for this approach as well. Guozhang On Thu, Aug 6, 2015 at 9:16 AM, Gwen Shapira g...@confluent.io wrote: I prefer same repo for one-commit / lower-barrier benefits. Sqoop has the following process, which decouples documentation changes from website changes: 1. Code github repo contains a doc directory, with the documentation written and maintained in AsciiDoc. Only one version of the documentation, since it is source controlled with the code. (unlike current SVN where we have directories per version) 2. Build process compiles the AsciiDoc to HTML and PDF 3. When releasing, we post the documentation of the new release to the website Gwen On Thu, Aug 6, 2015 at 12:20 AM, Ismael Juma ism...@juma.me.uk wrote: Hi, For reference, here is the previous discussion on moving the website to Git: http://search-hadoop.com/m/uyzND11JliU1E8QU92 People were positive to the idea as Jay said. I would like to see a bit of a discussion around whether the website should be part of the same repo as the code or not. I'll get the ball rolling. Pros for same repo: * One commit can update the code and website, which means: ** Lower barrier for updating docs along with relevant code changes ** Easier to require that both are updated at the same time * More eyeballs on the website changes * Automatically branched with the relevant code Pros for separate repo: * Potentially simpler for website-only changes (smaller repo, less verification needed) * Website changes don't clutter the code Git history * No risk of website change affecting the code Your thoughts, please. Best, Ismael On Fri, Jul 31, 2015 at 6:15 PM, Aseem Bansal asmbans...@gmail.com wrote: Hi When discussing on KAFKA-2364 migrating docs from svn to git came up. That would make contributing to docs much easier. I have contributed to groovy/grails via github so I think having mirror on github could be useful. Also I think unless there is some good reason it should be a separate repo. No need to mix docs and code. I can try that out. Thoughts? -- -- Guozhang -- Grant Henke Software Engineer | Cloudera gr...@cloudera.com | twitter.com/gchenke |
[jira] [Created] (KAFKA-2426) A Kafka node tries to connect to itself through its advertised hostname
Mikaël Cluseau created KAFKA-2426: - Summary: A Kafka node tries to connect to itself through its advertised hostname Key: KAFKA-2426 URL: https://issues.apache.org/jira/browse/KAFKA-2426 Project: Kafka Issue Type: Bug Components: network Affects Versions: 0.8.2.1 Environment: Docker https://github.com/wurstmeister/kafka-docker, managed by a Kubernetes cluster, with an iptables proxy. Reporter: Mikaël Cluseau Assignee: Jun Rao Hi, when used behind a firewall, Apache Kafka nodes are trying to connect to themselves using their advertised hostnames. This means that if you have a service IP managed by the docker's host using *only* iptables DNAT rules, the node's connection to itself times out. This is the case in any setup where a host will DNAT the service IP to the instance's IP, and send the packet back on the same interface other a Linux Bridge port not configured in hairpin mode. It's because of this: https://git.kernel.org/cgit/linux/kernel/git/torvalds/linux.git/tree/net/bridge/br_forward.c#n30 The specific part of the kubernetes issue is here: https://github.com/BenTheElder/kubernetes/issues/3#issuecomment-123925060 . The timeout involves that the even if partition's leader is elected, it then fails to accept writes from the other members, causing a write lock. and generating very heavy logs (as fast as Kafka usualy is, but through log4j this time ;)). This also means that the normal docker case work by going through the userspace-proxy, which necessarily impacts the performance. The workaround for us was to add a 127.0.0.2 {advertised hostname} to /etc/hosts in the container startup script. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Commented] (KAFKA-2425) Migrate website from SVN to Git
[ https://issues.apache.org/jira/browse/KAFKA-2425?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14693316#comment-14693316 ] Aseem Bansal commented on KAFKA-2425: - Yes I am interested. But how to do that. I mean I can take the checkout of kafka code from https://github.com/apache/kafka. Where can I get the SVN code? Also anything specific to take care of? Migrate website from SVN to Git Key: KAFKA-2425 URL: https://issues.apache.org/jira/browse/KAFKA-2425 Project: Kafka Issue Type: Improvement Reporter: Ismael Juma The preference is to share the same Git repo for the code and website as per discussion in the mailing list: http://search-hadoop.com/m/uyzND1Dux842dm7vg2 Useful reference: https://blogs.apache.org/infra/entry/git_based_websites_available -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Comment Edited] (KAFKA-2425) Migrate website from SVN to Git
[ https://issues.apache.org/jira/browse/KAFKA-2425?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14693330#comment-14693330 ] Manikumar Reddy edited comment on KAFKA-2425 at 8/12/15 11:24 AM: -- You can find the docs here : http://svn.apache.org/repos/asf/kafka/site Things to consider/to do: # Raise an Apache INFRA ticket to finalize the migration process. INFRA prefers new asf-site branch for site docs.. # checkout the site docs from svn and remove all .svn folders # open a github pull request and commit the site docs # Inform INFRA team to complete the migration. was (Author: omkreddy): You can find the docs here : http://svn.apache.org/repos/asf/kafka/site Things to consider/to do: # Raise an Apache INFRA ticket to finalize the migration process. INFRA prefers to new asf-site branch for site docs.. # checkout the site docs from svn and remove all .svn folders # open a github pull request and commit the site docs # Inform INFRA team to complete the migration. Migrate website from SVN to Git Key: KAFKA-2425 URL: https://issues.apache.org/jira/browse/KAFKA-2425 Project: Kafka Issue Type: Improvement Reporter: Ismael Juma The preference is to share the same Git repo for the code and website as per discussion in the mailing list: http://search-hadoop.com/m/uyzND1Dux842dm7vg2 Useful reference: https://blogs.apache.org/infra/entry/git_based_websites_available -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Commented] (KAFKA-2073) Replace TopicMetadata request/response with o.a.k.requests.metadata
[ https://issues.apache.org/jira/browse/KAFKA-2073?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14693859#comment-14693859 ] David Jacot commented on KAFKA-2073: [~abiletskyi] Are you still working on this one? Replace TopicMetadata request/response with o.a.k.requests.metadata --- Key: KAFKA-2073 URL: https://issues.apache.org/jira/browse/KAFKA-2073 Project: Kafka Issue Type: Sub-task Reporter: Gwen Shapira Assignee: Andrii Biletskyi Fix For: 0.8.3 Replace TopicMetadata request/response with o.a.k.requests.metadata. Note, this is more challenging that it appears because while the wire protocol is identical, the objects are completely different. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Commented] (KAFKA-2071) Replace Produce Request/Response with their org.apache.kafka.common.requests equivalents
[ https://issues.apache.org/jira/browse/KAFKA-2071?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14693855#comment-14693855 ] David Jacot commented on KAFKA-2071: [~gwenshap] Could you review this one when you have time? Replace Produce Request/Response with their org.apache.kafka.common.requests equivalents Key: KAFKA-2071 URL: https://issues.apache.org/jira/browse/KAFKA-2071 Project: Kafka Issue Type: Sub-task Reporter: Gwen Shapira Assignee: David Jacot Fix For: 0.8.3 Replace Produce Request/Response with their org.apache.kafka.common.requests equivalents -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Commented] (KAFKA-2425) Migrate website from SVN to Git
[ https://issues.apache.org/jira/browse/KAFKA-2425?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14693544#comment-14693544 ] Manikumar Reddy commented on KAFKA-2425: Issue Type can be simple Bug. SVN-GIT Migration is for migrating svn repo to git repo Sample Ticket: https://issues.apache.org/jira/browse/INFRA-10032 Migrate website from SVN to Git Key: KAFKA-2425 URL: https://issues.apache.org/jira/browse/KAFKA-2425 Project: Kafka Issue Type: Improvement Reporter: Ismael Juma The preference is to share the same Git repo for the code and website as per discussion in the mailing list: http://search-hadoop.com/m/uyzND1Dux842dm7vg2 Useful reference: https://blogs.apache.org/infra/entry/git_based_websites_available -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Comment Edited] (KAFKA-2425) Migrate website from SVN to Git
[ https://issues.apache.org/jira/browse/KAFKA-2425?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14693505#comment-14693505 ] Aseem Bansal edited comment on KAFKA-2425 at 8/12/15 1:54 PM: -- The Infra ticket has fields Git Notification Mailing List and Git Repository Import Path. I am not sure what they are. Project: Infrastructure Issue Type: SVN-GIT Migration was (Author: anshbansal): The Infra ticket has fields Git Notification Mailing List and Git Repository Import Path. I am not sure what they are. Migrate website from SVN to Git Key: KAFKA-2425 URL: https://issues.apache.org/jira/browse/KAFKA-2425 Project: Kafka Issue Type: Improvement Reporter: Ismael Juma The preference is to share the same Git repo for the code and website as per discussion in the mailing list: http://search-hadoop.com/m/uyzND1Dux842dm7vg2 Useful reference: https://blogs.apache.org/infra/entry/git_based_websites_available -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Commented] (KAFKA-2425) Migrate website from SVN to Git
[ https://issues.apache.org/jira/browse/KAFKA-2425?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14693505#comment-14693505 ] Aseem Bansal commented on KAFKA-2425: - The Infra ticket has fields Git Notification Mailing List and Git Repository Import Path. I am not sure what they are. Migrate website from SVN to Git Key: KAFKA-2425 URL: https://issues.apache.org/jira/browse/KAFKA-2425 Project: Kafka Issue Type: Improvement Reporter: Ismael Juma The preference is to share the same Git repo for the code and website as per discussion in the mailing list: http://search-hadoop.com/m/uyzND1Dux842dm7vg2 Useful reference: https://blogs.apache.org/infra/entry/git_based_websites_available -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Commented] (KAFKA-2418) Typo on official KAFKA documentation
[ https://issues.apache.org/jira/browse/KAFKA-2418?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14694010#comment-14694010 ] Gwen Shapira commented on KAFKA-2418: - Committed! Our docs look better now :) Thanks for fixing [~eribeiro]. Review took longer than expected since I had to re-learn SVN... Typo on official KAFKA documentation Key: KAFKA-2418 URL: https://issues.apache.org/jira/browse/KAFKA-2418 Project: Kafka Issue Type: Bug Components: website Affects Versions: 0.8.0, 0.8.1, 0.8.2.0 Reporter: Edward Ribeiro Assignee: Edward Ribeiro Priority: Trivial Attachments: KAFKA-2418.patch I have just seen the typo below at http://kafka.apache.org/documentation.html . By the end of the document there's a reference to JMZ instead of JMX. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Updated] (KAFKA-2418) Typo on official KAFKA documentation
[ https://issues.apache.org/jira/browse/KAFKA-2418?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Gwen Shapira updated KAFKA-2418: Resolution: Fixed Reviewer: Gwen Shapira Status: Resolved (was: Patch Available) Typo on official KAFKA documentation Key: KAFKA-2418 URL: https://issues.apache.org/jira/browse/KAFKA-2418 Project: Kafka Issue Type: Bug Components: website Affects Versions: 0.8.0, 0.8.1, 0.8.2.0 Reporter: Edward Ribeiro Assignee: Edward Ribeiro Priority: Trivial Attachments: KAFKA-2418.patch I have just seen the typo below at http://kafka.apache.org/documentation.html . By the end of the document there's a reference to JMZ instead of JMX. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
Re: Review Request 33049: Patch for KAFKA-2084
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/33049/ --- (Updated Aug. 12, 2015, 7:02 p.m.) Review request for kafka, Joel Koshy and Jun Rao. Bugs: KAFKA-2084 https://issues.apache.org/jira/browse/KAFKA-2084 Repository: kafka Description (updated) --- Signed-off-by: Aditya Auradkar aaurad...@linkedin.com Addressing Joel's comments Minor imports changes Added testcase to verify that replication traffic is not throttled Tmp commit Fixing test failure Minor Addressing Joel's comments Addressing comments Addressing comments Addressing Juns comments Minor checkstyle changes fixed test case Addressing Juns comments Addressing Juns comments Addressing Juns comments Diffs (updated) - clients/src/main/java/org/apache/kafka/common/metrics/Quota.java d82bb0c055e631425bc1ebbc7d387baac76aeeaa clients/src/main/java/org/apache/kafka/common/metrics/QuotaViolationException.java a451e5385c9eca76b38b425e8ac856b2715fcffe clients/src/main/java/org/apache/kafka/common/metrics/Sensor.java ca823fd4639523018311b814fde69b6177e73b97 clients/src/main/java/org/apache/kafka/common/metrics/stats/Rate.java 98429da34418f7f1deba1b5e44e2e6025212edb3 clients/src/test/java/org/apache/kafka/common/metrics/MetricsTest.java 544e120594de78c43581a980b1e4087b4fb98ccb clients/src/test/java/org/apache/kafka/common/utils/MockTime.java core/src/main/scala/kafka/server/ClientQuotaManager.scala PRE-CREATION core/src/main/scala/kafka/server/KafkaApis.scala 7ea509c2c41acc00430c74e025e069a833aac4e7 core/src/main/scala/kafka/server/KafkaConfig.scala dbe170f87331f43e2dc30165080d2cb7dfe5fdbf core/src/main/scala/kafka/server/KafkaServer.scala 84d4730ac634f9a5bf12a656e422fea03ad72da8 core/src/main/scala/kafka/server/ReplicaManager.scala 795220e7f63d163be90738b4c1a39687b44c1395 core/src/main/scala/kafka/server/ThrottledResponse.scala PRE-CREATION core/src/main/scala/kafka/utils/ShutdownableThread.scala fc226c863095b7761290292cd8755cd7ad0f155c core/src/test/scala/integration/kafka/api/QuotasTest.scala PRE-CREATION core/src/test/scala/unit/kafka/server/ClientQuotaManagerTest.scala PRE-CREATION core/src/test/scala/unit/kafka/server/KafkaConfigTest.scala f32d206d3f52f3f9f4d649c213edd7058f4b6150 core/src/test/scala/unit/kafka/server/ThrottledResponseExpirationTest.scala PRE-CREATION Diff: https://reviews.apache.org/r/33049/diff/ Testing --- Thanks, Aditya Auradkar
Re: Review Request 33049: Patch for KAFKA-2084
On Aug. 12, 2015, 12:34 a.m., Jun Rao wrote: clients/src/main/java/org/apache/kafka/common/metrics/Sensor.java, lines 131-150 https://reviews.apache.org/r/33049/diff/21/?file=1037164#file1037164line131 I think the comment can be a simpler. Basically, if O is the observed rate and T is the target rate over a window of W, to bring O down to T, we need to add a delay of X to W such that O * W / (W + X) = T. Solving for X, we get X = W*(O - T)/T. Thanks. Your comment is much better On Aug. 12, 2015, 12:34 a.m., Jun Rao wrote: clients/src/main/java/org/apache/kafka/common/metrics/Sensor.java, line 153 https://reviews.apache.org/r/33049/diff/21/?file=1037164#file1037164line153 Instead of using config.samples() * config.timeWindowMs(), shouldn't we use the formula elapsedCurrentWindowMs + elapsedPriorWindowsMs that we used in Rate.measure()? We can pass in now all the way from record(). Currently, the metric does not expose the underlying stat so I cannot do this computation in Sensor.java. The newer patch computes this in Rate.java and throws a QuotaViolationException from within. This way, we dont need to have a special comment in Sensor.java regarding rate metrics. - Aditya --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/33049/#review95033 --- On Aug. 11, 2015, 4:58 a.m., Aditya Auradkar wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/33049/ --- (Updated Aug. 11, 2015, 4:58 a.m.) Review request for kafka, Joel Koshy and Jun Rao. Bugs: KAFKA-2084 https://issues.apache.org/jira/browse/KAFKA-2084 Repository: kafka Description --- Updated patch for quotas. This patch does the following: 1. Add per-client metrics for both producer and consumers 2. Add configuration for quotas 3. Compute delay times in the metrics package and return the delay times in QuotaViolationException 4. Add a DelayQueue in KafkaApi's that can be used to throttle any type of request. Implemented request throttling for produce and fetch requests. 5. Added unit and integration test cases for both producer and consumer 6. This doesn't include a system test. There is a separate ticket for that 7. Fixed KAFKA-2191 - (Included fix from : https://reviews.apache.org/r/34418/ ) Addressed comments from Joel and Jun Diffs - clients/src/main/java/org/apache/kafka/common/metrics/Quota.java d82bb0c055e631425bc1ebbc7d387baac76aeeaa clients/src/main/java/org/apache/kafka/common/metrics/QuotaViolationException.java a451e5385c9eca76b38b425e8ac856b2715fcffe clients/src/main/java/org/apache/kafka/common/metrics/Sensor.java ca823fd4639523018311b814fde69b6177e73b97 clients/src/main/java/org/apache/kafka/common/metrics/stats/Rate.java 98429da34418f7f1deba1b5e44e2e6025212edb3 clients/src/test/java/org/apache/kafka/common/metrics/MetricsTest.java 544e120594de78c43581a980b1e4087b4fb98ccb clients/src/test/java/org/apache/kafka/common/utils/MockTime.java core/src/main/scala/kafka/server/ClientQuotaManager.scala PRE-CREATION core/src/main/scala/kafka/server/KafkaApis.scala 7ea509c2c41acc00430c74e025e069a833aac4e7 core/src/main/scala/kafka/server/KafkaConfig.scala dbe170f87331f43e2dc30165080d2cb7dfe5fdbf core/src/main/scala/kafka/server/KafkaServer.scala 84d4730ac634f9a5bf12a656e422fea03ad72da8 core/src/main/scala/kafka/server/ReplicaManager.scala 795220e7f63d163be90738b4c1a39687b44c1395 core/src/main/scala/kafka/server/ThrottledResponse.scala PRE-CREATION core/src/main/scala/kafka/utils/ShutdownableThread.scala fc226c863095b7761290292cd8755cd7ad0f155c core/src/test/scala/integration/kafka/api/QuotasTest.scala PRE-CREATION core/src/test/scala/unit/kafka/server/ClientQuotaManagerTest.scala PRE-CREATION core/src/test/scala/unit/kafka/server/KafkaConfigTest.scala f32d206d3f52f3f9f4d649c213edd7058f4b6150 core/src/test/scala/unit/kafka/server/ThrottledResponseExpirationTest.scala PRE-CREATION Diff: https://reviews.apache.org/r/33049/diff/ Testing --- Thanks, Aditya Auradkar
Re: Review Request 33049: Patch for KAFKA-2084
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/33049/ --- (Updated Aug. 12, 2015, 7:03 p.m.) Review request for kafka, Joel Koshy and Jun Rao. Bugs: KAFKA-2084 https://issues.apache.org/jira/browse/KAFKA-2084 Repository: kafka Description (updated) --- Updated patch for quotas. This patch does the following: 1. Add per-client metrics for both producer and consumers 2. Add configuration for quotas 3. Compute delay times in the metrics package and return the delay times in QuotaViolationException 4. Add a DelayQueue in KafkaApi's that can be used to throttle any type of request. Implemented request throttling for produce and fetch requests. 5. Added unit and integration test cases for both producer and consumer 6. This doesn't include a system test. There is a separate ticket for that 7. Fixed KAFKA-2191 - (Included fix from : https://reviews.apache.org/r/34418/ ) Addressed comments from Joel and Jun Diffs - clients/src/main/java/org/apache/kafka/common/metrics/Quota.java d82bb0c055e631425bc1ebbc7d387baac76aeeaa clients/src/main/java/org/apache/kafka/common/metrics/QuotaViolationException.java a451e5385c9eca76b38b425e8ac856b2715fcffe clients/src/main/java/org/apache/kafka/common/metrics/Sensor.java ca823fd4639523018311b814fde69b6177e73b97 clients/src/main/java/org/apache/kafka/common/metrics/stats/Rate.java 98429da34418f7f1deba1b5e44e2e6025212edb3 clients/src/test/java/org/apache/kafka/common/metrics/MetricsTest.java 544e120594de78c43581a980b1e4087b4fb98ccb clients/src/test/java/org/apache/kafka/common/utils/MockTime.java core/src/main/scala/kafka/server/ClientQuotaManager.scala PRE-CREATION core/src/main/scala/kafka/server/KafkaApis.scala 7ea509c2c41acc00430c74e025e069a833aac4e7 core/src/main/scala/kafka/server/KafkaConfig.scala dbe170f87331f43e2dc30165080d2cb7dfe5fdbf core/src/main/scala/kafka/server/KafkaServer.scala 84d4730ac634f9a5bf12a656e422fea03ad72da8 core/src/main/scala/kafka/server/ReplicaManager.scala 795220e7f63d163be90738b4c1a39687b44c1395 core/src/main/scala/kafka/server/ThrottledResponse.scala PRE-CREATION core/src/main/scala/kafka/utils/ShutdownableThread.scala fc226c863095b7761290292cd8755cd7ad0f155c core/src/test/scala/integration/kafka/api/QuotasTest.scala PRE-CREATION core/src/test/scala/unit/kafka/server/ClientQuotaManagerTest.scala PRE-CREATION core/src/test/scala/unit/kafka/server/KafkaConfigTest.scala f32d206d3f52f3f9f4d649c213edd7058f4b6150 core/src/test/scala/unit/kafka/server/ThrottledResponseExpirationTest.scala PRE-CREATION Diff: https://reviews.apache.org/r/33049/diff/ Testing --- Thanks, Aditya Auradkar
[jira] [Updated] (KAFKA-2084) byte rate metrics per client ID (producer and consumer)
[ https://issues.apache.org/jira/browse/KAFKA-2084?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Aditya A Auradkar updated KAFKA-2084: - Attachment: KAFKA-2084_2015-08-12_12:02:33.patch byte rate metrics per client ID (producer and consumer) --- Key: KAFKA-2084 URL: https://issues.apache.org/jira/browse/KAFKA-2084 Project: Kafka Issue Type: Sub-task Reporter: Aditya Auradkar Assignee: Aditya Auradkar Labels: quotas Attachments: KAFKA-2084.patch, KAFKA-2084_2015-04-09_18:10:56.patch, KAFKA-2084_2015-04-10_17:24:34.patch, KAFKA-2084_2015-04-21_12:21:18.patch, KAFKA-2084_2015-04-21_12:28:05.patch, KAFKA-2084_2015-05-05_15:27:35.patch, KAFKA-2084_2015-05-05_17:52:02.patch, KAFKA-2084_2015-05-11_16:16:01.patch, KAFKA-2084_2015-05-26_11:50:50.patch, KAFKA-2084_2015-06-02_17:02:00.patch, KAFKA-2084_2015-06-02_17:09:28.patch, KAFKA-2084_2015-06-02_17:10:52.patch, KAFKA-2084_2015-06-04_16:31:22.patch, KAFKA-2084_2015-06-12_10:39:35.patch, KAFKA-2084_2015-06-29_17:53:44.patch, KAFKA-2084_2015-08-04_18:50:51.patch, KAFKA-2084_2015-08-04_19:07:46.patch, KAFKA-2084_2015-08-07_11:27:51.patch, KAFKA-2084_2015-08-10_13:48:50.patch, KAFKA-2084_2015-08-10_21:57:48.patch, KAFKA-2084_2015-08-12_12:02:33.patch We need to be able to track the bytes-in/bytes-out rate on a per-client ID basis. This is necessary for quotas. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Commented] (KAFKA-2084) byte rate metrics per client ID (producer and consumer)
[ https://issues.apache.org/jira/browse/KAFKA-2084?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14694017#comment-14694017 ] Aditya A Auradkar commented on KAFKA-2084: -- Updated reviewboard https://reviews.apache.org/r/33049/diff/ against branch origin/trunk byte rate metrics per client ID (producer and consumer) --- Key: KAFKA-2084 URL: https://issues.apache.org/jira/browse/KAFKA-2084 Project: Kafka Issue Type: Sub-task Reporter: Aditya Auradkar Assignee: Aditya Auradkar Labels: quotas Attachments: KAFKA-2084.patch, KAFKA-2084_2015-04-09_18:10:56.patch, KAFKA-2084_2015-04-10_17:24:34.patch, KAFKA-2084_2015-04-21_12:21:18.patch, KAFKA-2084_2015-04-21_12:28:05.patch, KAFKA-2084_2015-05-05_15:27:35.patch, KAFKA-2084_2015-05-05_17:52:02.patch, KAFKA-2084_2015-05-11_16:16:01.patch, KAFKA-2084_2015-05-26_11:50:50.patch, KAFKA-2084_2015-06-02_17:02:00.patch, KAFKA-2084_2015-06-02_17:09:28.patch, KAFKA-2084_2015-06-02_17:10:52.patch, KAFKA-2084_2015-06-04_16:31:22.patch, KAFKA-2084_2015-06-12_10:39:35.patch, KAFKA-2084_2015-06-29_17:53:44.patch, KAFKA-2084_2015-08-04_18:50:51.patch, KAFKA-2084_2015-08-04_19:07:46.patch, KAFKA-2084_2015-08-07_11:27:51.patch, KAFKA-2084_2015-08-10_13:48:50.patch, KAFKA-2084_2015-08-10_21:57:48.patch, KAFKA-2084_2015-08-12_12:02:33.patch We need to be able to track the bytes-in/bytes-out rate on a per-client ID basis. This is necessary for quotas. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
Re: Review Request 33049: Patch for KAFKA-2084
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/33049/ --- (Updated Aug. 12, 2015, 7:04 p.m.) Review request for kafka, Joel Koshy and Jun Rao. Bugs: KAFKA-2084 https://issues.apache.org/jira/browse/KAFKA-2084 Repository: kafka Description (updated) --- Signed-off-by: Aditya Auradkar aaurad...@linkedin.com Addressing Joel's comments Minor imports changes Added testcase to verify that replication traffic is not throttled Tmp commit Fixing test failure Minor Addressing Joel's comments Addressing comments Addressing comments Addressing Juns comments Minor checkstyle changes fixed test case Addressing Juns comments Addressing Juns comments Addressing Juns comments Minor change Diffs (updated) - clients/src/main/java/org/apache/kafka/common/metrics/Quota.java d82bb0c055e631425bc1ebbc7d387baac76aeeaa clients/src/main/java/org/apache/kafka/common/metrics/QuotaViolationException.java a451e5385c9eca76b38b425e8ac856b2715fcffe clients/src/main/java/org/apache/kafka/common/metrics/Sensor.java ca823fd4639523018311b814fde69b6177e73b97 clients/src/main/java/org/apache/kafka/common/metrics/stats/Rate.java 98429da34418f7f1deba1b5e44e2e6025212edb3 clients/src/test/java/org/apache/kafka/common/metrics/MetricsTest.java 544e120594de78c43581a980b1e4087b4fb98ccb clients/src/test/java/org/apache/kafka/common/utils/MockTime.java core/src/main/scala/kafka/server/ClientQuotaManager.scala PRE-CREATION core/src/main/scala/kafka/server/KafkaApis.scala 7ea509c2c41acc00430c74e025e069a833aac4e7 core/src/main/scala/kafka/server/KafkaConfig.scala dbe170f87331f43e2dc30165080d2cb7dfe5fdbf core/src/main/scala/kafka/server/KafkaServer.scala 84d4730ac634f9a5bf12a656e422fea03ad72da8 core/src/main/scala/kafka/server/ReplicaManager.scala 795220e7f63d163be90738b4c1a39687b44c1395 core/src/main/scala/kafka/server/ThrottledResponse.scala PRE-CREATION core/src/main/scala/kafka/utils/ShutdownableThread.scala fc226c863095b7761290292cd8755cd7ad0f155c core/src/test/scala/integration/kafka/api/QuotasTest.scala PRE-CREATION core/src/test/scala/unit/kafka/server/ClientQuotaManagerTest.scala PRE-CREATION core/src/test/scala/unit/kafka/server/KafkaConfigTest.scala f32d206d3f52f3f9f4d649c213edd7058f4b6150 core/src/test/scala/unit/kafka/server/ThrottledResponseExpirationTest.scala PRE-CREATION Diff: https://reviews.apache.org/r/33049/diff/ Testing --- Thanks, Aditya Auradkar
[jira] [Updated] (KAFKA-2084) byte rate metrics per client ID (producer and consumer)
[ https://issues.apache.org/jira/browse/KAFKA-2084?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Aditya A Auradkar updated KAFKA-2084: - Attachment: KAFKA-2084_2015-08-12_12:04:51.patch byte rate metrics per client ID (producer and consumer) --- Key: KAFKA-2084 URL: https://issues.apache.org/jira/browse/KAFKA-2084 Project: Kafka Issue Type: Sub-task Reporter: Aditya Auradkar Assignee: Aditya Auradkar Labels: quotas Attachments: KAFKA-2084.patch, KAFKA-2084_2015-04-09_18:10:56.patch, KAFKA-2084_2015-04-10_17:24:34.patch, KAFKA-2084_2015-04-21_12:21:18.patch, KAFKA-2084_2015-04-21_12:28:05.patch, KAFKA-2084_2015-05-05_15:27:35.patch, KAFKA-2084_2015-05-05_17:52:02.patch, KAFKA-2084_2015-05-11_16:16:01.patch, KAFKA-2084_2015-05-26_11:50:50.patch, KAFKA-2084_2015-06-02_17:02:00.patch, KAFKA-2084_2015-06-02_17:09:28.patch, KAFKA-2084_2015-06-02_17:10:52.patch, KAFKA-2084_2015-06-04_16:31:22.patch, KAFKA-2084_2015-06-12_10:39:35.patch, KAFKA-2084_2015-06-29_17:53:44.patch, KAFKA-2084_2015-08-04_18:50:51.patch, KAFKA-2084_2015-08-04_19:07:46.patch, KAFKA-2084_2015-08-07_11:27:51.patch, KAFKA-2084_2015-08-10_13:48:50.patch, KAFKA-2084_2015-08-10_21:57:48.patch, KAFKA-2084_2015-08-12_12:02:33.patch, KAFKA-2084_2015-08-12_12:04:51.patch We need to be able to track the bytes-in/bytes-out rate on a per-client ID basis. This is necessary for quotas. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Commented] (KAFKA-2084) byte rate metrics per client ID (producer and consumer)
[ https://issues.apache.org/jira/browse/KAFKA-2084?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14694021#comment-14694021 ] Aditya A Auradkar commented on KAFKA-2084: -- Updated reviewboard https://reviews.apache.org/r/33049/diff/ against branch origin/trunk byte rate metrics per client ID (producer and consumer) --- Key: KAFKA-2084 URL: https://issues.apache.org/jira/browse/KAFKA-2084 Project: Kafka Issue Type: Sub-task Reporter: Aditya Auradkar Assignee: Aditya Auradkar Labels: quotas Attachments: KAFKA-2084.patch, KAFKA-2084_2015-04-09_18:10:56.patch, KAFKA-2084_2015-04-10_17:24:34.patch, KAFKA-2084_2015-04-21_12:21:18.patch, KAFKA-2084_2015-04-21_12:28:05.patch, KAFKA-2084_2015-05-05_15:27:35.patch, KAFKA-2084_2015-05-05_17:52:02.patch, KAFKA-2084_2015-05-11_16:16:01.patch, KAFKA-2084_2015-05-26_11:50:50.patch, KAFKA-2084_2015-06-02_17:02:00.patch, KAFKA-2084_2015-06-02_17:09:28.patch, KAFKA-2084_2015-06-02_17:10:52.patch, KAFKA-2084_2015-06-04_16:31:22.patch, KAFKA-2084_2015-06-12_10:39:35.patch, KAFKA-2084_2015-06-29_17:53:44.patch, KAFKA-2084_2015-08-04_18:50:51.patch, KAFKA-2084_2015-08-04_19:07:46.patch, KAFKA-2084_2015-08-07_11:27:51.patch, KAFKA-2084_2015-08-10_13:48:50.patch, KAFKA-2084_2015-08-10_21:57:48.patch, KAFKA-2084_2015-08-12_12:02:33.patch, KAFKA-2084_2015-08-12_12:04:51.patch We need to be able to track the bytes-in/bytes-out rate on a per-client ID basis. This is necessary for quotas. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
Re: Review Request 33049: Patch for KAFKA-2084
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/33049/ --- (Updated Aug. 12, 2015, 7:05 p.m.) Review request for kafka, Joel Koshy and Jun Rao. Bugs: KAFKA-2084 https://issues.apache.org/jira/browse/KAFKA-2084 Repository: kafka Description (updated) --- Updated patch for quotas. This patch does the following: 1. Add per-client metrics for both producer and consumers 2. Add configuration for quotas 3. Compute delay times in the metrics package and return the delay times in QuotaViolationException 4. Add a DelayQueue in KafkaApi's that can be used to throttle any type of request. Implemented request throttling for produce and fetch requests. 5. Added unit and integration test cases for both producer and consumer 6. This doesn't include a system test. There is a separate ticket for that 7. Fixed KAFKA-2191 - (Included fix from : https://reviews.apache.org/r/34418/ ) Addressed comments from Joel and Jun Diffs - clients/src/main/java/org/apache/kafka/common/metrics/Quota.java d82bb0c055e631425bc1ebbc7d387baac76aeeaa clients/src/main/java/org/apache/kafka/common/metrics/QuotaViolationException.java a451e5385c9eca76b38b425e8ac856b2715fcffe clients/src/main/java/org/apache/kafka/common/metrics/Sensor.java ca823fd4639523018311b814fde69b6177e73b97 clients/src/main/java/org/apache/kafka/common/metrics/stats/Rate.java 98429da34418f7f1deba1b5e44e2e6025212edb3 clients/src/test/java/org/apache/kafka/common/metrics/MetricsTest.java 544e120594de78c43581a980b1e4087b4fb98ccb clients/src/test/java/org/apache/kafka/common/utils/MockTime.java core/src/main/scala/kafka/server/ClientQuotaManager.scala PRE-CREATION core/src/main/scala/kafka/server/KafkaApis.scala 7ea509c2c41acc00430c74e025e069a833aac4e7 core/src/main/scala/kafka/server/KafkaConfig.scala dbe170f87331f43e2dc30165080d2cb7dfe5fdbf core/src/main/scala/kafka/server/KafkaServer.scala 84d4730ac634f9a5bf12a656e422fea03ad72da8 core/src/main/scala/kafka/server/ReplicaManager.scala 795220e7f63d163be90738b4c1a39687b44c1395 core/src/main/scala/kafka/server/ThrottledResponse.scala PRE-CREATION core/src/main/scala/kafka/utils/ShutdownableThread.scala fc226c863095b7761290292cd8755cd7ad0f155c core/src/test/scala/integration/kafka/api/QuotasTest.scala PRE-CREATION core/src/test/scala/unit/kafka/server/ClientQuotaManagerTest.scala PRE-CREATION core/src/test/scala/unit/kafka/server/KafkaConfigTest.scala f32d206d3f52f3f9f4d649c213edd7058f4b6150 core/src/test/scala/unit/kafka/server/ThrottledResponseExpirationTest.scala PRE-CREATION Diff: https://reviews.apache.org/r/33049/diff/ Testing --- Thanks, Aditya Auradkar
Re: Review Request 33049: Patch for KAFKA-2084
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/33049/ --- (Updated Aug. 12, 2015, 7:08 p.m.) Review request for kafka, Joel Koshy and Jun Rao. Bugs: KAFKA-2084 https://issues.apache.org/jira/browse/KAFKA-2084 Repository: kafka Description (updated) --- Signed-off-by: Aditya Auradkar aaurad...@linkedin.com Addressing Joel's comments Minor imports changes Added testcase to verify that replication traffic is not throttled Tmp commit Fixing test failure Minor Addressing Joel's comments Addressing comments Addressing comments Addressing Juns comments Minor checkstyle changes fixed test case Addressing Juns comments Addressing Juns comments Addressing Juns comments Minor change Minor change Diffs (updated) - clients/src/main/java/org/apache/kafka/common/metrics/Quota.java d82bb0c055e631425bc1ebbc7d387baac76aeeaa clients/src/main/java/org/apache/kafka/common/metrics/QuotaViolationException.java a451e5385c9eca76b38b425e8ac856b2715fcffe clients/src/main/java/org/apache/kafka/common/metrics/Sensor.java ca823fd4639523018311b814fde69b6177e73b97 clients/src/main/java/org/apache/kafka/common/metrics/stats/Rate.java 98429da34418f7f1deba1b5e44e2e6025212edb3 clients/src/test/java/org/apache/kafka/common/metrics/MetricsTest.java 544e120594de78c43581a980b1e4087b4fb98ccb clients/src/test/java/org/apache/kafka/common/utils/MockTime.java core/src/main/scala/kafka/server/ClientQuotaManager.scala PRE-CREATION core/src/main/scala/kafka/server/KafkaApis.scala 7ea509c2c41acc00430c74e025e069a833aac4e7 core/src/main/scala/kafka/server/KafkaConfig.scala dbe170f87331f43e2dc30165080d2cb7dfe5fdbf core/src/main/scala/kafka/server/KafkaServer.scala 84d4730ac634f9a5bf12a656e422fea03ad72da8 core/src/main/scala/kafka/server/ReplicaManager.scala 795220e7f63d163be90738b4c1a39687b44c1395 core/src/main/scala/kafka/server/ThrottledResponse.scala PRE-CREATION core/src/main/scala/kafka/utils/ShutdownableThread.scala fc226c863095b7761290292cd8755cd7ad0f155c core/src/test/scala/integration/kafka/api/QuotasTest.scala PRE-CREATION core/src/test/scala/unit/kafka/server/ClientQuotaManagerTest.scala PRE-CREATION core/src/test/scala/unit/kafka/server/KafkaConfigTest.scala f32d206d3f52f3f9f4d649c213edd7058f4b6150 core/src/test/scala/unit/kafka/server/ThrottledResponseExpirationTest.scala PRE-CREATION Diff: https://reviews.apache.org/r/33049/diff/ Testing --- Thanks, Aditya Auradkar
[jira] [Updated] (KAFKA-2084) byte rate metrics per client ID (producer and consumer)
[ https://issues.apache.org/jira/browse/KAFKA-2084?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Aditya A Auradkar updated KAFKA-2084: - Attachment: KAFKA-2084_2015-08-12_12:08:17.patch byte rate metrics per client ID (producer and consumer) --- Key: KAFKA-2084 URL: https://issues.apache.org/jira/browse/KAFKA-2084 Project: Kafka Issue Type: Sub-task Reporter: Aditya Auradkar Assignee: Aditya Auradkar Labels: quotas Attachments: KAFKA-2084.patch, KAFKA-2084_2015-04-09_18:10:56.patch, KAFKA-2084_2015-04-10_17:24:34.patch, KAFKA-2084_2015-04-21_12:21:18.patch, KAFKA-2084_2015-04-21_12:28:05.patch, KAFKA-2084_2015-05-05_15:27:35.patch, KAFKA-2084_2015-05-05_17:52:02.patch, KAFKA-2084_2015-05-11_16:16:01.patch, KAFKA-2084_2015-05-26_11:50:50.patch, KAFKA-2084_2015-06-02_17:02:00.patch, KAFKA-2084_2015-06-02_17:09:28.patch, KAFKA-2084_2015-06-02_17:10:52.patch, KAFKA-2084_2015-06-04_16:31:22.patch, KAFKA-2084_2015-06-12_10:39:35.patch, KAFKA-2084_2015-06-29_17:53:44.patch, KAFKA-2084_2015-08-04_18:50:51.patch, KAFKA-2084_2015-08-04_19:07:46.patch, KAFKA-2084_2015-08-07_11:27:51.patch, KAFKA-2084_2015-08-10_13:48:50.patch, KAFKA-2084_2015-08-10_21:57:48.patch, KAFKA-2084_2015-08-12_12:02:33.patch, KAFKA-2084_2015-08-12_12:04:51.patch, KAFKA-2084_2015-08-12_12:08:17.patch We need to be able to track the bytes-in/bytes-out rate on a per-client ID basis. This is necessary for quotas. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Commented] (KAFKA-2084) byte rate metrics per client ID (producer and consumer)
[ https://issues.apache.org/jira/browse/KAFKA-2084?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14694023#comment-14694023 ] Aditya A Auradkar commented on KAFKA-2084: -- Updated reviewboard https://reviews.apache.org/r/33049/diff/ against branch origin/trunk byte rate metrics per client ID (producer and consumer) --- Key: KAFKA-2084 URL: https://issues.apache.org/jira/browse/KAFKA-2084 Project: Kafka Issue Type: Sub-task Reporter: Aditya Auradkar Assignee: Aditya Auradkar Labels: quotas Attachments: KAFKA-2084.patch, KAFKA-2084_2015-04-09_18:10:56.patch, KAFKA-2084_2015-04-10_17:24:34.patch, KAFKA-2084_2015-04-21_12:21:18.patch, KAFKA-2084_2015-04-21_12:28:05.patch, KAFKA-2084_2015-05-05_15:27:35.patch, KAFKA-2084_2015-05-05_17:52:02.patch, KAFKA-2084_2015-05-11_16:16:01.patch, KAFKA-2084_2015-05-26_11:50:50.patch, KAFKA-2084_2015-06-02_17:02:00.patch, KAFKA-2084_2015-06-02_17:09:28.patch, KAFKA-2084_2015-06-02_17:10:52.patch, KAFKA-2084_2015-06-04_16:31:22.patch, KAFKA-2084_2015-06-12_10:39:35.patch, KAFKA-2084_2015-06-29_17:53:44.patch, KAFKA-2084_2015-08-04_18:50:51.patch, KAFKA-2084_2015-08-04_19:07:46.patch, KAFKA-2084_2015-08-07_11:27:51.patch, KAFKA-2084_2015-08-10_13:48:50.patch, KAFKA-2084_2015-08-10_21:57:48.patch, KAFKA-2084_2015-08-12_12:02:33.patch, KAFKA-2084_2015-08-12_12:04:51.patch, KAFKA-2084_2015-08-12_12:08:17.patch We need to be able to track the bytes-in/bytes-out rate on a per-client ID basis. This is necessary for quotas. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
Re: Review Request 33049: Patch for KAFKA-2084
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/33049/ --- (Updated Aug. 12, 2015, 7:09 p.m.) Review request for kafka, Joel Koshy and Jun Rao. Bugs: KAFKA-2084 https://issues.apache.org/jira/browse/KAFKA-2084 Repository: kafka Description (updated) --- Updated patch for quotas. This patch does the following: 1. Add per-client metrics for both producer and consumers 2. Add configuration for quotas 3. Compute delay times in the metrics package and return the delay times in QuotaViolationException 4. Add a DelayQueue in KafkaApi's that can be used to throttle any type of request. Implemented request throttling for produce and fetch requests. 5. Added unit and integration test cases for both producer and consumer 6. This doesn't include a system test. There is a separate ticket for that 7. Fixed KAFKA-2191 - (Included fix from : https://reviews.apache.org/r/34418/ ) Addressed comments from Joel and Jun Diffs - clients/src/main/java/org/apache/kafka/common/metrics/Quota.java d82bb0c055e631425bc1ebbc7d387baac76aeeaa clients/src/main/java/org/apache/kafka/common/metrics/QuotaViolationException.java a451e5385c9eca76b38b425e8ac856b2715fcffe clients/src/main/java/org/apache/kafka/common/metrics/Sensor.java ca823fd4639523018311b814fde69b6177e73b97 clients/src/main/java/org/apache/kafka/common/metrics/stats/Rate.java 98429da34418f7f1deba1b5e44e2e6025212edb3 clients/src/test/java/org/apache/kafka/common/metrics/MetricsTest.java 544e120594de78c43581a980b1e4087b4fb98ccb clients/src/test/java/org/apache/kafka/common/utils/MockTime.java core/src/main/scala/kafka/server/ClientQuotaManager.scala PRE-CREATION core/src/main/scala/kafka/server/KafkaApis.scala 7ea509c2c41acc00430c74e025e069a833aac4e7 core/src/main/scala/kafka/server/KafkaConfig.scala dbe170f87331f43e2dc30165080d2cb7dfe5fdbf core/src/main/scala/kafka/server/KafkaServer.scala 84d4730ac634f9a5bf12a656e422fea03ad72da8 core/src/main/scala/kafka/server/ReplicaManager.scala 795220e7f63d163be90738b4c1a39687b44c1395 core/src/main/scala/kafka/server/ThrottledResponse.scala PRE-CREATION core/src/main/scala/kafka/utils/ShutdownableThread.scala fc226c863095b7761290292cd8755cd7ad0f155c core/src/test/scala/integration/kafka/api/QuotasTest.scala PRE-CREATION core/src/test/scala/unit/kafka/server/ClientQuotaManagerTest.scala PRE-CREATION core/src/test/scala/unit/kafka/server/KafkaConfigTest.scala f32d206d3f52f3f9f4d649c213edd7058f4b6150 core/src/test/scala/unit/kafka/server/ThrottledResponseExpirationTest.scala PRE-CREATION Diff: https://reviews.apache.org/r/33049/diff/ Testing --- Thanks, Aditya Auradkar
[jira] [Commented] (KAFKA-2425) Migrate website from SVN to Git
[ https://issues.apache.org/jira/browse/KAFKA-2425?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14694034#comment-14694034 ] Gwen Shapira commented on KAFKA-2425: - Wouldn't a new branch mean that we can't push code and doc changes together, so the benefits of using git (or at least same repo) are reduced? Migrate website from SVN to Git Key: KAFKA-2425 URL: https://issues.apache.org/jira/browse/KAFKA-2425 Project: Kafka Issue Type: Improvement Reporter: Ismael Juma The preference is to share the same Git repo for the code and website as per discussion in the mailing list: http://search-hadoop.com/m/uyzND1Dux842dm7vg2 Useful reference: https://blogs.apache.org/infra/entry/git_based_websites_available -- This message was sent by Atlassian JIRA (v6.3.4#6332)
Re: Review Request 33049: Patch for KAFKA-2084
On Aug. 12, 2015, 12:42 a.m., Jun Rao wrote: core/src/main/scala/kafka/server/KafkaConfig.scala, line 419 https://reviews.apache.org/r/33049/diff/21/?file=1037170#file1037170line419 I am still not sure that I see the value of the delay factor. If one wants to be a bit conservative, one can always configure a lower quota value. I'm dropping this config since it is quite unintuitive. Other reviewers have also mentioned this - Aditya --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/33049/#review95035 --- On Aug. 11, 2015, 4:58 a.m., Aditya Auradkar wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/33049/ --- (Updated Aug. 11, 2015, 4:58 a.m.) Review request for kafka, Joel Koshy and Jun Rao. Bugs: KAFKA-2084 https://issues.apache.org/jira/browse/KAFKA-2084 Repository: kafka Description --- Updated patch for quotas. This patch does the following: 1. Add per-client metrics for both producer and consumers 2. Add configuration for quotas 3. Compute delay times in the metrics package and return the delay times in QuotaViolationException 4. Add a DelayQueue in KafkaApi's that can be used to throttle any type of request. Implemented request throttling for produce and fetch requests. 5. Added unit and integration test cases for both producer and consumer 6. This doesn't include a system test. There is a separate ticket for that 7. Fixed KAFKA-2191 - (Included fix from : https://reviews.apache.org/r/34418/ ) Addressed comments from Joel and Jun Diffs - clients/src/main/java/org/apache/kafka/common/metrics/Quota.java d82bb0c055e631425bc1ebbc7d387baac76aeeaa clients/src/main/java/org/apache/kafka/common/metrics/QuotaViolationException.java a451e5385c9eca76b38b425e8ac856b2715fcffe clients/src/main/java/org/apache/kafka/common/metrics/Sensor.java ca823fd4639523018311b814fde69b6177e73b97 clients/src/main/java/org/apache/kafka/common/metrics/stats/Rate.java 98429da34418f7f1deba1b5e44e2e6025212edb3 clients/src/test/java/org/apache/kafka/common/metrics/MetricsTest.java 544e120594de78c43581a980b1e4087b4fb98ccb clients/src/test/java/org/apache/kafka/common/utils/MockTime.java core/src/main/scala/kafka/server/ClientQuotaManager.scala PRE-CREATION core/src/main/scala/kafka/server/KafkaApis.scala 7ea509c2c41acc00430c74e025e069a833aac4e7 core/src/main/scala/kafka/server/KafkaConfig.scala dbe170f87331f43e2dc30165080d2cb7dfe5fdbf core/src/main/scala/kafka/server/KafkaServer.scala 84d4730ac634f9a5bf12a656e422fea03ad72da8 core/src/main/scala/kafka/server/ReplicaManager.scala 795220e7f63d163be90738b4c1a39687b44c1395 core/src/main/scala/kafka/server/ThrottledResponse.scala PRE-CREATION core/src/main/scala/kafka/utils/ShutdownableThread.scala fc226c863095b7761290292cd8755cd7ad0f155c core/src/test/scala/integration/kafka/api/QuotasTest.scala PRE-CREATION core/src/test/scala/unit/kafka/server/ClientQuotaManagerTest.scala PRE-CREATION core/src/test/scala/unit/kafka/server/KafkaConfigTest.scala f32d206d3f52f3f9f4d649c213edd7058f4b6150 core/src/test/scala/unit/kafka/server/ThrottledResponseExpirationTest.scala PRE-CREATION Diff: https://reviews.apache.org/r/33049/diff/ Testing --- Thanks, Aditya Auradkar
[jira] [Updated] (KAFKA-2120) Add a request timeout to NetworkClient
[ https://issues.apache.org/jira/browse/KAFKA-2120?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Mayuresh Gharat updated KAFKA-2120: --- Attachment: KAFKA-2120_2015-08-12_10:59:09.patch Add a request timeout to NetworkClient -- Key: KAFKA-2120 URL: https://issues.apache.org/jira/browse/KAFKA-2120 Project: Kafka Issue Type: New Feature Reporter: Jiangjie Qin Assignee: Mayuresh Gharat Fix For: 0.8.3 Attachments: KAFKA-2120.patch, KAFKA-2120_2015-07-27_15:31:19.patch, KAFKA-2120_2015-07-29_15:57:02.patch, KAFKA-2120_2015-08-10_19:55:18.patch, KAFKA-2120_2015-08-12_10:59:09.patch Currently NetworkClient does not have a timeout setting for requests. So if no response is received for a request due to reasons such as broker is down, the request will never be completed. Request timeout will also be used as implicit timeout for some methods such as KafkaProducer.flush() and kafkaProducer.close(). KIP-19 is created for this public interface change. https://cwiki.apache.org/confluence/display/KAFKA/KIP-19+-+Add+a+request+timeout+to+NetworkClient -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Commented] (KAFKA-2120) Add a request timeout to NetworkClient
[ https://issues.apache.org/jira/browse/KAFKA-2120?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14693951#comment-14693951 ] Mayuresh Gharat commented on KAFKA-2120: Updated reviewboard https://reviews.apache.org/r/36858/diff/ against branch origin/trunk Add a request timeout to NetworkClient -- Key: KAFKA-2120 URL: https://issues.apache.org/jira/browse/KAFKA-2120 Project: Kafka Issue Type: New Feature Reporter: Jiangjie Qin Assignee: Mayuresh Gharat Fix For: 0.8.3 Attachments: KAFKA-2120.patch, KAFKA-2120_2015-07-27_15:31:19.patch, KAFKA-2120_2015-07-29_15:57:02.patch, KAFKA-2120_2015-08-10_19:55:18.patch, KAFKA-2120_2015-08-12_10:59:09.patch Currently NetworkClient does not have a timeout setting for requests. So if no response is received for a request due to reasons such as broker is down, the request will never be completed. Request timeout will also be used as implicit timeout for some methods such as KafkaProducer.flush() and kafkaProducer.close(). KIP-19 is created for this public interface change. https://cwiki.apache.org/confluence/display/KAFKA/KIP-19+-+Add+a+request+timeout+to+NetworkClient -- This message was sent by Atlassian JIRA (v6.3.4#6332)
Re: [DISCUSS] Client-side Assignment for New Consumer
Hey Ewen, Onur and I discussed this a little bit more. And we are still worrying about passing all the metadata of all consumers around. Let's say I have a cluster has 10,000 topics, the average topic name length is 10 bytes. In this case, the opaque metadata will have 10 * 10,000 = 100KB for topic name, for each topic, there is a 4-byte integer of number of partitions, that's another 40KB. So one global topic metadata will have 140KB data. If I have 100 consumers who are using wildcard to consume from all the topics. That means the protocol metadata end up in the JoinGroupResponse will be 140KB * 100 = 14MB. And the JoinGroupResponse will need to be sent to 100 different consumers, that means 14MB * 100 = 1.4GB need to be sent by the consumer coordinator for one rebalance. How would that work? Also, having two consumers (old owner and new owner) consuming from the same partition might also be a problem. e.g. people are updating database. One thing might worth doing is to add GroupId and Generation ID to ProducerRequest and FetchRequest as well. This will also help with the single producer use case. However, this is probably orthogonal to this thread given the current new consumer also has this problem and I believe we need to fix it. Thanks, Jiangjie (Becket) Qin On Tue, Aug 11, 2015 at 11:43 PM, Ewen Cheslack-Postava e...@confluent.io wrote: On Tue, Aug 11, 2015 at 11:29 PM, Jiangjie Qin j...@linkedin.com.invalid wrote: Ewen, Thanks for the explanation. For (1), I am more concerned about the failure case instead of normal case. What if a consumer somehow was kick out of a group but is still consuming and committing offsets? Does that mean the new owner and old owner might potentially consuming from and committing offsets for the same partition? In the old consumer, this won't happen because the new consumer will not be able to start consumption unless the previous owner has released its ownership. Basically, without the ownership guarantee, I don't see how the communication among consumers themselves alone can solve the problem here. The generation ID check still applies to offset commits. If one of the consumers is kicked out and misbehaving, it can obviously still fetch and process messages, but offset commits will not work since it will not have the current generation ID. For (2) and (3), now I understand how metadata are used. But I still don't see why should we let the consumers to pass the topic information across instead of letting coordinator give the information. The single producer use case does not solve the ownership problem in abnormal case either, which seems to be a little bit vulnerable. One of the goals here was to generalize group membership so we can, for example, use it for balancing Copycat tasks across workers. There's no topic subscription info in that case. The metadata for copycat workers would instead need to somehow indicate the current set of tasks that need to be assigned to workers. By making the metadata completely opaque to the protocol, it becomes more generally useful since it focuses squarely on the group membership problem, allowing for that additional bit of metadata so you don't just get a list of members, but also get a little bit of info about each of them. A different option that we explored is to use a sort of mixed model -- still bake all the topic subscriptions directly into the protocol but also include metadata. That would allow us to maintain the existing coordinator-driven approach to handling the metadata and change events like the ones Onur pointed out. Then something like the Copycat workers would just not fill in any topic subscriptions and it would be handled as a degenerate case. Based on the way I explained that we can handle those types of events, I personally feel its cleaner and a nicer generalization to not include the subscriptions in the join group protocol, making it part of the metadata instead. For the single producer case, are you saying it doesn't solve ownership in the abnormal case because a producer that doesn't know it has been kicked out of the group yet can still produce data even though it shouldn't be able to anymore? I definitely agree that that is a risk -- this provides a way to get closer to a true single-writer, but there are definitely still failure modes that this does not address. -Ewen Thanks, Jiangjie (Becket) Qin On Tue, Aug 11, 2015 at 11:06 PM, Ewen Cheslack-Postava e...@confluent.io wrote: On Tue, Aug 11, 2015 at 10:15 PM, Jiangjie Qin j...@linkedin.com.invalid wrote: Hi Jason, Thanks for writing this up. It would be useful to generalize the group concept. I have a few questions below. 1. In old consumer actually the partition assignment are done by consumers themselves. We used zookeeper to guarantee that a partition will only be consumed by one
[jira] [Assigned] (KAFKA-2015) Enable ConsoleConsumer to use new consumer
[ https://issues.apache.org/jira/browse/KAFKA-2015?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Ben Stopford reassigned KAFKA-2015: --- Assignee: Ben Stopford (was: Guozhang Wang) Enable ConsoleConsumer to use new consumer -- Key: KAFKA-2015 URL: https://issues.apache.org/jira/browse/KAFKA-2015 Project: Kafka Issue Type: Sub-task Components: consumer Reporter: Guozhang Wang Assignee: Ben Stopford Fix For: 0.9.0 Attachments: KAFKA-2015.patch As titled, enable ConsoleConsumer to use new consumer. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
Re: Kafka Indentation
+1 about rewriting to Java (in a year or more). +1 about upping Scala use in the meanwhile +1 about using Gradle instead of Maven or SBT. Eddie On Wed, Aug 12, 2015 at 5:26 AM, Ismael Juma ism...@juma.me.uk wrote: On Wed, Aug 12, 2015 at 1:29 AM, Gwen Shapira g...@confluent.io wrote: +1 on not breaking git blame -1 on rewriting Kafka in Java +1 on upping our Scala game (as Ismael pointed out) I filed a couple of JIRAs, and I'll look at introducing Scalastyle once some of the bigger patches/PRs are merged. https://issues.apache.org/jira/browse/KAFKA-2423 https://issues.apache.org/jira/browse/KAFKA-2424 Regarding the rewrite of Kafka in Java, I think it's a bit early to be voting on that given that Jay's suggestion was to do it once we can use Java 8 in the broker (over a year away). I'm with you Gwen, let's up our Scala game in the meantime. Once the time comes to make the decision, we can evaluate the pros and cons and have the vote. Ismael
[jira] [Commented] (KAFKA-2418) Typo on official KAFKA documentation
[ https://issues.apache.org/jira/browse/KAFKA-2418?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14693984#comment-14693984 ] Edward Ribeiro commented on KAFKA-2418: --- Hi folks, if you have any free time could someone review this and push it to trunk, please? :) I didn't setup a reviewer yet because I know committers and contributors are super busy with coding/reviewing more important stuff. Cheers! /cc [~junrao] [~gwenshap] ? Typo on official KAFKA documentation Key: KAFKA-2418 URL: https://issues.apache.org/jira/browse/KAFKA-2418 Project: Kafka Issue Type: Bug Components: website Affects Versions: 0.8.0, 0.8.1, 0.8.2.0 Reporter: Edward Ribeiro Assignee: Edward Ribeiro Priority: Trivial Attachments: KAFKA-2418.patch I have just seen the typo below at http://kafka.apache.org/documentation.html . By the end of the document there's a reference to JMZ instead of JMX. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Assigned] (KAFKA-2072) Add StopReplica request/response to o.a.k.common.requests and replace the usage in core module
[ https://issues.apache.org/jira/browse/KAFKA-2072?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] David Jacot reassigned KAFKA-2072: -- Assignee: David Jacot Add StopReplica request/response to o.a.k.common.requests and replace the usage in core module -- Key: KAFKA-2072 URL: https://issues.apache.org/jira/browse/KAFKA-2072 Project: Kafka Issue Type: Sub-task Reporter: Gwen Shapira Assignee: David Jacot Fix For: 0.8.3 Add StopReplica request/response to o.a.k.common.requests and replace the usage in core module -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Created] (KAFKA-2428) Add sanity test in kafkaConsumer for the timeouts. This is a followup ticket for Kafka-2120
Mayuresh Gharat created KAFKA-2428: -- Summary: Add sanity test in kafkaConsumer for the timeouts. This is a followup ticket for Kafka-2120 Key: KAFKA-2428 URL: https://issues.apache.org/jira/browse/KAFKA-2428 Project: Kafka Issue Type: Bug Reporter: Mayuresh Gharat Assignee: Mayuresh Gharat The request timeout should be the highest timeout across all the timeout. The rules should be: Request timeout session timeout. Request timeout fetch.max.wait.timeout request timeout won't kick in before the other timeout reached. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
Re: Review Request 36858: Patch for KAFKA-2120
On Aug. 11, 2015, 8:49 p.m., Jason Gustafson wrote: clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerConfig.java, line 302 https://reviews.apache.org/r/36858/diff/4/?file=1037078#file1037078line302 Can we make this value greater than sessionTimeoutMs (which is 30s). Even if we don't address the issue of sanity between the different timeouts in this patch, it would be nice to have compatible defaults to keep the consumer from breaking out of the box. Mayuresh Gharat wrote: Hi Jason, I will upload a new patch with the sanity test. Hi Jason, I am creating a new jira ticket for the sanity test. This is because the requestTimeout in KafkaConsumer is long but it has to be int. This is because it is used as timeout in produceRequest that is used as replicationTimeout on the server side. The change might occur at multiple places and is completely orthogonal to this KIP. Here is the jira ticket : https://issues.apache.org/jira/browse/KAFKA-2428 - Mayuresh --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/36858/#review94999 --- On Aug. 11, 2015, 2:55 a.m., Mayuresh Gharat wrote: --- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/36858/ --- (Updated Aug. 11, 2015, 2:55 a.m.) Review request for kafka. Bugs: KAFKA-2120 https://issues.apache.org/jira/browse/KAFKA-2120 Repository: kafka Description --- Solved compile error Addressed Jason's comments for Kip-19 Addressed Jun's comments Diffs - clients/src/main/java/org/apache/kafka/clients/ClientRequest.java dc8f0f115bcda893c95d17c0a57be8d14518d034 clients/src/main/java/org/apache/kafka/clients/CommonClientConfigs.java 2c421f42ed3fc5d61cf9c87a7eaa7bb23e26f63b clients/src/main/java/org/apache/kafka/clients/InFlightRequests.java 15d00d4e484bb5d51a9ae6857ed6e024a2cc1820 clients/src/main/java/org/apache/kafka/clients/KafkaClient.java 7ab2503794ff3aab39df881bd9fbae6547827d3b clients/src/main/java/org/apache/kafka/clients/NetworkClient.java 0e51d7bd461d253f4396a5b6ca7cd391658807fa clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerConfig.java d35b421a515074d964c7fccb73d260b847ea5f00 clients/src/main/java/org/apache/kafka/clients/consumer/KafkaConsumer.java ed99e9bdf7c4ea7a6d4555d4488cf8ed0b80641b clients/src/main/java/org/apache/kafka/clients/consumer/internals/ConsumerNetworkClient.java 9517d9d0cd480d5ba1d12f1fde7963e60528d2f8 clients/src/main/java/org/apache/kafka/clients/producer/KafkaProducer.java 03b8dd23df63a8d8a117f02eabcce4a2d48c44f7 clients/src/main/java/org/apache/kafka/clients/producer/ProducerConfig.java aa264202f2724907924985a5ecbe74afc4c6c04b clients/src/main/java/org/apache/kafka/clients/producer/internals/BufferPool.java 4cb1e50d6c4ed55241aeaef1d3af09def5274103 clients/src/main/java/org/apache/kafka/clients/producer/internals/RecordAccumulator.java a152bd7697dca55609a9ec4cfe0a82c10595fbc3 clients/src/main/java/org/apache/kafka/clients/producer/internals/RecordBatch.java 06182db1c3a5da85648199b4c0c98b80ea7c6c0c clients/src/main/java/org/apache/kafka/clients/producer/internals/Sender.java 0baf16e55046a2f49f6431e01d52c323c95eddf0 clients/src/main/java/org/apache/kafka/common/network/Selector.java ce20111ac434eb8c74585e9c63757bb9d60a832f clients/src/test/java/org/apache/kafka/clients/MockClient.java 9133d85342b11ba2c9888d4d2804d181831e7a8e clients/src/test/java/org/apache/kafka/clients/NetworkClientTest.java 43238ceaad0322e39802b615bb805b895336a009 clients/src/test/java/org/apache/kafka/clients/producer/internals/BufferPoolTest.java 2c693824fa53db1e38766b8c66a0ef42ef9d0f3a clients/src/test/java/org/apache/kafka/clients/producer/internals/RecordAccumulatorTest.java 5b2e4ffaeab7127648db608c179703b27b577414 clients/src/test/java/org/apache/kafka/clients/producer/internals/SenderTest.java 8b1805d3d2bcb9fe2bacb37d870c3236aa9532c4 clients/src/test/java/org/apache/kafka/common/network/SelectorTest.java 158f9829ff64a969008f699e40c51e918287859e core/src/main/scala/kafka/tools/ProducerPerformance.scala 0335cc64013ffe2cdf1c4879e86e11ec8c526712 core/src/test/scala/integration/kafka/api/ProducerFailureHandlingTest.scala ee94011894b46864614b97bbd2a98375a7d3f20b core/src/test/scala/unit/kafka/utils/TestUtils.scala eb169d8b33c27d598cc24e5a2e5f78b789fa38d3 Diff: https://reviews.apache.org/r/36858/diff/ Testing --- Thanks, Mayuresh Gharat
[jira] [Commented] (KAFKA-2071) Replace Produce Request/Response with their org.apache.kafka.common.requests equivalents
[ https://issues.apache.org/jira/browse/KAFKA-2071?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanelfocusedCommentId=14693874#comment-14693874 ] Gwen Shapira commented on KAFKA-2071: - Its on my review queue, so I definitely will. It may take me few days to get to it. Replace Produce Request/Response with their org.apache.kafka.common.requests equivalents Key: KAFKA-2071 URL: https://issues.apache.org/jira/browse/KAFKA-2071 Project: Kafka Issue Type: Sub-task Reporter: Gwen Shapira Assignee: David Jacot Fix For: 0.8.3 Replace Produce Request/Response with their org.apache.kafka.common.requests equivalents -- This message was sent by Atlassian JIRA (v6.3.4#6332)
Re: Review Request 36858: Patch for KAFKA-2120
--- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/36858/ --- (Updated Aug. 12, 2015, 5:59 p.m.) Review request for kafka. Bugs: KAFKA-2120 https://issues.apache.org/jira/browse/KAFKA-2120 Repository: kafka Description (updated) --- Solved compile error Addressed Jason's comments for Kip-19 Addressed Jun's comments Addressed Jason's comments about the default values for requestTimeout Diffs (updated) - clients/src/main/java/org/apache/kafka/clients/ClientRequest.java dc8f0f115bcda893c95d17c0a57be8d14518d034 clients/src/main/java/org/apache/kafka/clients/CommonClientConfigs.java 2c421f42ed3fc5d61cf9c87a7eaa7bb23e26f63b clients/src/main/java/org/apache/kafka/clients/InFlightRequests.java 15d00d4e484bb5d51a9ae6857ed6e024a2cc1820 clients/src/main/java/org/apache/kafka/clients/KafkaClient.java 7ab2503794ff3aab39df881bd9fbae6547827d3b clients/src/main/java/org/apache/kafka/clients/NetworkClient.java 0e51d7bd461d253f4396a5b6ca7cd391658807fa clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerConfig.java d35b421a515074d964c7fccb73d260b847ea5f00 clients/src/main/java/org/apache/kafka/clients/consumer/KafkaConsumer.java ed99e9bdf7c4ea7a6d4555d4488cf8ed0b80641b clients/src/main/java/org/apache/kafka/clients/consumer/internals/ConsumerNetworkClient.java 9517d9d0cd480d5ba1d12f1fde7963e60528d2f8 clients/src/main/java/org/apache/kafka/clients/producer/KafkaProducer.java 03b8dd23df63a8d8a117f02eabcce4a2d48c44f7 clients/src/main/java/org/apache/kafka/clients/producer/ProducerConfig.java aa264202f2724907924985a5ecbe74afc4c6c04b clients/src/main/java/org/apache/kafka/clients/producer/internals/BufferPool.java 4cb1e50d6c4ed55241aeaef1d3af09def5274103 clients/src/main/java/org/apache/kafka/clients/producer/internals/RecordAccumulator.java a152bd7697dca55609a9ec4cfe0a82c10595fbc3 clients/src/main/java/org/apache/kafka/clients/producer/internals/RecordBatch.java 06182db1c3a5da85648199b4c0c98b80ea7c6c0c clients/src/main/java/org/apache/kafka/clients/producer/internals/Sender.java 0baf16e55046a2f49f6431e01d52c323c95eddf0 clients/src/main/java/org/apache/kafka/common/network/Selector.java ce20111ac434eb8c74585e9c63757bb9d60a832f clients/src/test/java/org/apache/kafka/clients/MockClient.java 9133d85342b11ba2c9888d4d2804d181831e7a8e clients/src/test/java/org/apache/kafka/clients/NetworkClientTest.java 43238ceaad0322e39802b615bb805b895336a009 clients/src/test/java/org/apache/kafka/clients/producer/internals/BufferPoolTest.java 2c693824fa53db1e38766b8c66a0ef42ef9d0f3a clients/src/test/java/org/apache/kafka/clients/producer/internals/RecordAccumulatorTest.java 5b2e4ffaeab7127648db608c179703b27b577414 clients/src/test/java/org/apache/kafka/clients/producer/internals/SenderTest.java 8b1805d3d2bcb9fe2bacb37d870c3236aa9532c4 clients/src/test/java/org/apache/kafka/common/network/SelectorTest.java 158f9829ff64a969008f699e40c51e918287859e core/src/main/scala/kafka/tools/ProducerPerformance.scala 0335cc64013ffe2cdf1c4879e86e11ec8c526712 core/src/test/scala/integration/kafka/api/ProducerFailureHandlingTest.scala ee94011894b46864614b97bbd2a98375a7d3f20b core/src/test/scala/unit/kafka/utils/TestUtils.scala eb169d8b33c27d598cc24e5a2e5f78b789fa38d3 Diff: https://reviews.apache.org/r/36858/diff/ Testing --- Thanks, Mayuresh Gharat