Skip to content

KAFKA-7471: Multiple Consumer Group Management Feature#5726

Merged
vahidhashemian merged 43 commits intoapache:trunkfrom
rootex-:KAFKA-7471_Multiple_Consumer_Group_Management_Feature
Apr 15, 2019
Merged

KAFKA-7471: Multiple Consumer Group Management Feature#5726
vahidhashemian merged 43 commits intoapache:trunkfrom
rootex-:KAFKA-7471_Multiple_Consumer_Group_Management_Feature

Conversation

@rootex-
Copy link
Copy Markdown
Contributor

@rootex- rootex- commented Oct 2, 2018

FEATURE
KAFKA-7471 Multiple Consumer Group Management (Describe, Reset, Delete) + --groups-all option
KIP-379: Multiple Consumer Group Management
[DISCUSS]: KIP-379: Multiple Consumer Group Management

Description:

  • Describe/Delete/Reset offsets on multiple consumer groups at a time (including each group by repeating --group parameter)
  • Describe/Delete/Reset offsets on ALL consumer groups at a time (add new --all-groups option similar to --all-topics)
  • Reset plan CSV file generation reworked: structure updated to support multiple consumer groups and make sure that CSV file generation is done properly since there are no restrictions on consumer group names and symbols like commas and quotes are allowed.
  • Extending data output table format by adding "GROUP" column for all --describe queries

All Unit and integration tests implemented.

Committer Checklist (excluded from commit message)

  • Verify design and implementation
  • Verify test coverage and CI build status
  • Verify documentation (including upgrade notes)

Alexander Alexandrovich Dunayevsky and others added 14 commits December 26, 2017 15:52
…e) + --groups-all option

Functionality:
* Describe/Delete/Reset offsets on multiple consumer groups at a time (including each group by repeating `--group` parameter)
* Describe/Delete/Reset offsets on ALL consumer groups at a time (add new `--groups-all` option similar to `--topics-all`)
* Export offsets to reset for multiple consumer groups to a CSV file (CSV generation export/import rework, CSV format rework)
@rootex- rootex- changed the title Kafka 7471 multiple consumer group management feature KAFKA-7471 Multiple Consumer Group Management Feature Oct 2, 2018
@rootex- rootex- changed the title KAFKA-7471 Multiple Consumer Group Management Feature KAFKA-7471: Multiple Consumer Group Management Feature Oct 2, 2018
rootex- and others added 2 commits October 3, 2018 17:24
…of github.com:rootex-/kafka into KAFKA-7471_Multiple_Consumer_Group_Management_Feature
@hachikuji
Copy link
Copy Markdown
Contributor

@rootex- Thanks a lot for the contribution. I think this is a great idea. There are a few comments/suggestions in the KIP discussion thread. Can you have a look and let us know what you think?

@rootex-
Copy link
Copy Markdown
Contributor Author

rootex- commented Oct 31, 2018

@rootex- Thanks a lot for the contribution. I think this is a great idea. There are a few comments/suggestions in the KIP discussion thread. Can you have a look and let us know what you think?

@hachikuji Hi Jason, thanks for messaging here. Missed the discussion thread without any email notifications. I'll check them out and leave my reply as soon as I can.

@saagar2000
Copy link
Copy Markdown

@Rootex @hachikuji Can all consumer groups functionality extended to admin client ? i.e change listConsumeGroupOffsets to allow take multiple group ids and return all consumer group offsets or have listAllConsumerGroupOffsets method.

@hachikuji
Copy link
Copy Markdown
Contributor

@saagar2000 Yeah, that's a good question It is a little curious that we made describeConsumerGroups a batched API, but we did not do the same for listConsumeGroupOffsets. I think the underlying reason is that the OffsetFetch API is not batched. We could consider changing it, of course. One awkward aspect of all the consumer group APIs is finding the right coordinators. So if you request offsets for two groups, you may end up sending two requests anyway because the coordinators may be on different brokers.

alex.dunayevsky and others added 4 commits January 8, 2019 22:16
@rootex-
Copy link
Copy Markdown
Contributor Author

rootex- commented Jan 10, 2019

Please review the latest updates, according to [DISCUSS]: KIP-379: Multiple Consumer Group Management

Copy link
Copy Markdown
Contributor

@vahidhashemian vahidhashemian left a comment

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

@rootex- Thank you for submitting the PR. I left a few initial comments.
BTW, the KIP has enough binding votes for approval.

Comment thread core/src/main/scala/kafka/admin/ConsumerGroupCommand.scala Outdated
Comment thread core/src/main/scala/kafka/admin/ConsumerGroupCommand.scala Outdated
Comment thread core/src/test/scala/unit/kafka/admin/DeleteConsumerGroupsTest.scala Outdated
Comment thread core/src/main/scala/kafka/admin/ConsumerGroupCommand.scala
@rootex-
Copy link
Copy Markdown
Contributor Author

rootex- commented Jan 25, 2019

  • one blank line is skipped.

Thanks, you're right once again. Previously, that preceding \n blank line before the Error message wasn't necessary since we expected --describe results from a single group. I also think it's a good idea.

  • The other cosmetic issue (misaligned column values) ...

Done, thanks! Check out, please

  • In the below example the last line seems to be unnecessary

I'll be back to this in a while

Copy link
Copy Markdown
Contributor

@vahidhashemian vahidhashemian left a comment

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Thanks for addressing my earlier comments. I left a comment inline, regarding one of those comments.

Comment thread core/src/main/scala/kafka/admin/ConsumerGroupCommand.scala Outdated
@vahidhashemian
Copy link
Copy Markdown
Contributor

Looks like there are build failures.

@rootex-
Copy link
Copy Markdown
Contributor Author

rootex- commented Feb 2, 2019

@vahidhashemian Yes, there were lots of conflicts and tricky merge. Looks like this branch is too far away from the original trunk version causing troubles. I'll take time to take a closer look.

alex.dunayevsky added 2 commits March 7, 2019 15:17
* Single group CSV format: "topic,partition,offset"
* Multiple group CSV format: "group,topic,partition,offset"
@rootex- rootex- force-pushed the KAFKA-7471_Multiple_Consumer_Group_Management_Feature branch from a15861b to 9bb66ff Compare March 11, 2019 12:19
@rootex-
Copy link
Copy Markdown
Contributor Author

rootex- commented Mar 11, 2019

Looks like there are build failures.

Back to work. Fixed now 👌

What's going on with trunk branch? It looks broken and does not even compile 😯

@rootex-
Copy link
Copy Markdown
Contributor Author

rootex- commented Mar 12, 2019

@vahidhashemian any further comments maybe?

@rootex-
Copy link
Copy Markdown
Contributor Author

rootex- commented Mar 21, 2019

@vahidhashemian @hachikuji any further notes or corrections maybe? It's ready to be merged I believe 👌

@rootex-
Copy link
Copy Markdown
Contributor Author

rootex- commented Mar 28, 2019

@vahidhashemian @hachikuji guys?

@vahidhashemian
Copy link
Copy Markdown
Contributor

I'm on vacation right now. Will try to take a look within the next week or so.

Copy link
Copy Markdown
Contributor

@vahidhashemian vahidhashemian left a comment

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Thanks for updating the PR and sorry for the delay in my review. I think it's close now. Left a few minor comments inline.

Comment thread core/src/main/scala/kafka/admin/ConsumerGroupCommand.scala Outdated
Comment thread core/src/main/scala/kafka/admin/ConsumerGroupCommand.scala Outdated
Comment thread core/src/main/scala/kafka/admin/ConsumerGroupCommand.scala Outdated
Comment thread core/src/main/scala/kafka/admin/ConsumerGroupCommand.scala Outdated
// Make sure we got a coordinator
TestUtils.waitUntilTrue(() => {
consumerGroupCommand.collectGroupState().coordinator.host() == "localhost"
}, "Can't find a coordinator
Copy link
Copy Markdown
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Was there a coordinator lookup issue that warranted this block? I couldn't get this test to fail without it.

Copy link
Copy Markdown
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Do not have any idea really, this code appeared after merge from trunk

Copy link
Copy Markdown
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Looks like the code has been committed by Gwen Shapira here to fix a test:
KAFKA-7937: Fix Flaky Test ResetConsumerGroupOffsetTest.testResetOffsetsNotExistingGroup (#6311)

Copy link
Copy Markdown
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

@vahidhashemian is it ok if we leave it as is?

Copy link
Copy Markdown
Contributor Author

@rootex- rootex- Apr 8, 2019

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

@vahidhashemian one more thing: I'm unable to compile my branch anymore after merging with trunk due to a bunch of missing classes in imports section of org.apache.kafka.common.protocol.ApiKeys.java class like:

import org.apache.kafka.common.message.CreateTopicsRequestData;
import org.apache.kafka.common.message.CreateTopicsResponseData;
import org.apache.kafka.common.message.DescribeGroupsRequestData;
import org.apache.kafka.common.message.DescribeGroupsResponseData;
import org.apache.kafka.common.message.ElectPreferredLeadersRequestData;
import org.apache.kafka.common.message.ElectPreferredLeadersResponseData;
import org.apache.kafka.common.message.LeaveGroupRequestData;
import org.apache.kafka.common.message.LeaveGroupResponseData;
import org.apache.kafka.common.message.MetadataRequestData;
import org.apache.kafka.common.message.MetadataResponseData;
import org.apache.kafka.common.message.SaslAuthenticateRequestData;
import org.apache.kafka.common.message.SaslAuthenticateResponseData;
import org.apache.kafka.common.message.SaslHandshakeRequestData;
import org.apache.kafka.common.message.SaslHandshakeResponseData;

Those files are missing in the project.
And I'm also unable to compile trunk as well for the same reason.

Could we fix that?

Copy link
Copy Markdown
Contributor

@vahidhashemian vahidhashemian Apr 8, 2019

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Regarding the added block of code I though you added it since it appeared in the commit diff. I now remember Gwen adding that block, so all is good. Sorry for the confusion.

I think I also noticed the compile issues you are referring to. Will have to dig further to determine the root cause.

Otherwise, things look good to me. Thanks for removing the unused imports.

Copy link
Copy Markdown
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

@vahidhashemian thanks for your accurate reviews, Vahid 👍 very straightforward and to the point

@rootex-
Copy link
Copy Markdown
Contributor Author

rootex- commented Apr 10, 2019

@hachikuji @vahidhashemian according to Vahid, PR is ready to be merged. All improvements have been applied. Can't wait to start using it out-of-the-box 😄 Jason, your final word?

Copy link
Copy Markdown
Contributor

@vahidhashemian vahidhashemian left a comment

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

@hachikuji Did you also want to take a look at this PR before merging it?

@hachikuji
Copy link
Copy Markdown
Contributor

@vahidhashemian I did a high level pass and it seems reasonable. If you're happy with it, I'd suggest merging.

Comment thread build.gradle
compile project(':clients')
compile libs.jacksonDatabind
compile libs.jacksonModuleScala
compile libs.jacksonDataformatCsv
Copy link
Copy Markdown
Member

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

@vahidhashemian I don't think we should have this dependency. We have avoided the Jackson module for Scala on purpose as it doesn't provide enough value. It would be good to fix this before 2.3.0 if at all possible. cc @hachikuji

Copy link
Copy Markdown
Member

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

To clarify, I'm referring to the Scala module (I got the line slightly wrong). The CSV one is probably fine.

Copy link
Copy Markdown
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

How this was isolated:

  • comment out libs.jacksonModuleScala dependency in build.gradle
  • execute ./gradlew -PscalaVersion=2.12 jar

Result: two compile errors in kafka.admin.ConsumerGroupCommand (core module)

I volunteer to help with this.

Copy link
Copy Markdown
Member

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

@dejan2609 If you submit a PR, I'll help review it.

Copy link
Copy Markdown
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

@ijuma Thanks for catching this. I wasn't aware of the limitation.
@dejan2609 Thanks for volunteering. If you like, you can open a JIRA for easier tracking of the issue.

Copy link
Copy Markdown
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

@vahidhashemian Sure, I will open JIRA ticket.

@ijuma FYI: as a former Java developer that switched to build/release engineering I can't deliver quickly (but I assume this is not a post-haste).

Copy link
Copy Markdown
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

JIRA ticket is created here: https://issues.apache.org/jira/browse/KAFKA-8466 Remove 'jackson-module-scala' dependency (and replace it with some code)

pengxiaolong pushed a commit to pengxiaolong/kafka that referenced this pull request Jun 14, 2019
* Describe/Delete/Reset offsets on multiple consumer groups at a time (including each group by repeating `--group` parameter)
* Describe/Delete/Reset offsets on ALL consumer groups at a time (add new `--all-groups` option similar to `--all-topics`)
* Reset plan CSV file generation reworked: structure updated to support multiple consumer groups and make sure that CSV file generation is done properly since there are no restrictions on consumer group names and symbols like commas and quotes are allowed.
* Extending data output table format by adding `GROUP` column for all `--describe` queries
Sign up for free to join this conversation on GitHub. Already have an account? Sign in to comment

Labels

None yet

Projects

None yet

Development

Successfully merging this pull request may close these issues.

6 participants