Skip to content
New issue

Have a question about this project? Sign up for a free GitHub account to open an issue and contact its maintainers and the community.

By clicking “Sign up for GitHub”, you agree to our terms of service and privacy statement. We’ll occasionally send you account related emails.

Already on GitHub? Sign in to your account

KAFKA-10028: Implement write path for feature versioning system (KIP-584) #9001

Merged
merged 41 commits into from
Oct 7, 2020

Conversation

kowshik
Copy link
Contributor

@kowshik kowshik commented Jul 9, 2020

Summary:
In this PR, I have implemented the write path of the feature versioning system (KIP-584). Here is a summary of what's in this PR:

  • New APIs in org.apache.kafka.clients.admin.Admin interface, and their client and server implementations. These APIs can be used to describe features and update finalized features. These APIs are: Admin#describeFeatures and Admin#updateFeatures.
    • The write path is provided by the Admin#updateFeatures API. The corresponding server-side implementation is provided in KafkaApis and KafkaController classes. This can be a good place to start the code review.
    • The write path is supplemented by Admin#describeFeatures client API. This does not translate 1:1 to a server-side API. Instead, under the hood the API makes an explicit ApiVersionsRequest to the Broker to fetch the supported and finalized features.
  • Implemented a suite of integration tests in UpdateFeaturesTest.scala that thoroughly exercises the various cases in the write path.

Other changes:

  • The data type of the FinalizedFeaturesEpoch field in ApiVersionsResponse has been modified from int32 to int64. This change is to conform with the latest changes to the KIP explained in the voting thread (see this link).
  • Along the way, the class SupportedFeatures has been renamed to be called BrokerFeatures, and, it now holds both supported features as well as default minimum version levels.
  • For the purpose of testing, both the BrokerFeatures and FinalizedFeatureCache classes have been changed to be no longer singleton in implementation. Instead, these are now instantiated once and maintained in KafkaServer. The singleton instances are passed around to various classes, as needed.

Test plan:

  • Relying on existing and unit and integration tests added in this PR.

Copy link
Contributor Author

@kowshik kowshik left a comment

Choose a reason for hiding this comment

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

First pass

@kowshik kowshik force-pushed the kip584_features_write_path branch 3 times, most recently from 8f6add3 to aa4cc4a Compare July 10, 2020 09:04
@dajac
Copy link
Member

dajac commented Jul 10, 2020

@kowshik I just noticed that you haven't updated the code which creates the ApiVersionsResponse in SaslServerAuthenticator. Is it intentional or something left to be done?

@kowshik kowshik force-pushed the kip584_features_write_path branch from aa4cc4a to 5370dd7 Compare July 10, 2020 21:30
@kowshik
Copy link
Contributor Author

kowshik commented Jul 10, 2020

@dajac Thank you for taking a look! IIUC you are referring to these lines:

https://github.com/apache/kafka/blob/trunk/clients/src/main/java/org/apache/kafka/common/security/authenticator/SaslServerAuthenticator.java#L550-L553

My requirement is that under the hood of the newly added API: org.apache.kafka.clients.Admin#describeFeatures, the ApiVersionsResponse returned to the AdminClient needs to contain the features information. Note that this new API issues an explicit ApiVersionsRequest under the hood. In such a case do you think I should populate the features information in the above lines in SaslServerAuthenticator too? I'm trying to understand where would this come into play (sorry I know little to nothing about SaslServerAuthenticator).

@kowshik kowshik force-pushed the kip584_features_write_path branch 9 times, most recently from c63313e to 88970cf Compare July 11, 2020 05:46
@kowshik kowshik changed the title KAFKA-10028: Implement KIP-584 write path KAFKA-10028: Implement write path for feature versioning system (KIP-584) Jul 11, 2020
@kowshik kowshik marked this pull request as ready for review July 11, 2020 05:58
@abbccdda abbccdda added the core Kafka Broker label Jul 11, 2020
@kowshik kowshik force-pushed the kip584_features_write_path branch 11 times, most recently from 1183b0e to 0eca1b9 Compare July 13, 2020 09:18
@kowshik kowshik force-pushed the kip584_features_write_path branch from e55358f to e1c79ce Compare October 6, 2020 22:58
@kowshik
Copy link
Contributor Author

kowshik commented Oct 6, 2020

@junrao The test failure in MirrorConnectorsIntegrationTest.testReplication does not seem related. I have rebased the PR now against latest AK trunk, I'd like to see if the failure happens again.

Copy link
Contributor

@abbccdda abbccdda left a comment

Choose a reason for hiding this comment

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

LGTM, thanks for the great work!

@kowshik
Copy link
Contributor Author

kowshik commented Oct 7, 2020

The test failures in the latest CI runs do not seem related to this PR:

  • JDK 8: org.apache.kafka.connect.integration.ExampleConnectIntegrationTest.testSourceConnector
  • JDK 11: org.apache.kafka.streams.integration.EosBetaUpgradeIntegrationTest.shouldUpgradeFromEosAlphaToEosBeta

The test that failed previously under JDK 15 has passed in the latest CI run: MirrorConnectorsIntegrationTest.testReplication.

@kowshik
Copy link
Contributor Author

kowshik commented Oct 7, 2020

@junrao Thanks for the links! I had a look at the links. I found similar stats in both links, with exactly 27 test failures in both links. I compared the individual test failures and found that they have all failed on the same tests. Would that mean we are OK to merge this PR (since it doesn't seem to introduce a new failure)?

@junrao
Copy link
Contributor

junrao commented Oct 7, 2020

@kowshik : Thanks for following up. I will merged this PR as it is since the system test failures are not new.

Also, in Scala, we prefer sealed traits over Enumeration since the former gives you exhaustiveness checking. With Scala enums, you don't get a warning if you add a new value that is not handled in a given pattern match. Maybe you can address that in your followup PR.

@junrao junrao merged commit fb4f297 into apache:trunk Oct 7, 2020
@chia7712
Copy link
Member

chia7712 commented Oct 8, 2020

@kowshik sorry for bringing trivial comments after this is merged. I just noticed those nits in testing new APIs in 2.7.0.

@kowshik
Copy link
Contributor Author

kowshik commented Oct 8, 2020

@chia7712 No worries, thanks for the suggestions! I have opened a separate PR addressing your comments. Would you be able to please review it? #9393

javierfreire pushed a commit to javierfreire/kafka that referenced this pull request Oct 8, 2020
…584) (apache#9001)

Summary:
In this PR, I have implemented the write path of the feature versioning system (KIP-584). Here is a summary of what's in this PR:

New APIs in org.apache.kafka.clients.admin.Admin interface, and their client and server implementations. These APIs can be used to describe features and update finalized features. These APIs are: Admin#describeFeatures and Admin#updateFeatures.
The write path is provided by the Admin#updateFeatures API. The corresponding server-side implementation is provided in KafkaApis and KafkaController classes. This can be a good place to start the code review.
The write path is supplemented by Admin#describeFeatures client API. This does not translate 1:1 to a server-side API. Instead, under the hood the API makes an explicit ApiVersionsRequest to the Broker to fetch the supported and finalized features.
Implemented a suite of integration tests in UpdateFeaturesTest.scala that thoroughly exercises the various cases in the write path.

Other changes:

The data type of the FinalizedFeaturesEpoch field in ApiVersionsResponse has been modified from int32 to int64. This change is to conform with the latest changes to the KIP explained in the voting thread.
Along the way, the class SupportedFeatures has been renamed to be called BrokerFeatures, and, it now holds both supported features as well as default minimum version levels.
For the purpose of testing, both the BrokerFeatures and FinalizedFeatureCache classes have been changed to be no longer singleton in implementation. Instead, these are now instantiated once and maintained in KafkaServer. The singleton instances are passed around to various classes, as needed.

Reviewers: Boyang Chen <[email protected]>, Jun Rao <[email protected]>
junrao pushed a commit that referenced this pull request Oct 8, 2020
…9393)

In this PR, I have addressed the review comments from @chia7712 in #9001 which were provided after #9001 was merged. The changes are made mainly to KafkaAdminClient:

Improve error message in updateFeatures api when feature name is empty.
Propagate top-level error message in updateFeatures api.
Add an empty-parameter variety for describeFeatures api.
Minor documentation updates to @param and @return to make these resemble other apis.

Reviewers: Chia-Ping Tsai [email protected], Jun Rao [email protected]
junrao pushed a commit that referenced this pull request Oct 8, 2020
…9393)

In this PR, I have addressed the review comments from @chia7712 in #9001 which were provided after #9001 was merged. The changes are made mainly to KafkaAdminClient:

Improve error message in updateFeatures api when feature name is empty.
Propagate top-level error message in updateFeatures api.
Add an empty-parameter variety for describeFeatures api.
Minor documentation updates to @param and @return to make these resemble other apis.

Reviewers: Chia-Ping Tsai [email protected], Jun Rao [email protected]
ijuma added a commit to confluentinc/kafka that referenced this pull request Oct 8, 2020
* commit '2804257fe221f37e5098bd': (67 commits)
  KAFKA-10562: Properly invoke new StateStoreContext init (apache#9388)
  MINOR: trivial cleanups, javadoc errors, omitted StateStore tests, etc. (apache#8130)
  KAFKA-10564: only process non-empty task directories when internally cleaning obsolete state stores (apache#9373)
  KAFKA-9274: fix incorrect default value for `task.timeout.ms` config (apache#9385)
  KAFKA-10362: When resuming Streams active task with EOS, the checkpoint file is deleted (apache#9247)
  KAFKA-10028: Implement write path for feature versioning system (KIP-584) (apache#9001)
  KAFKA-10402: Upgrade system tests to python3 (apache#9196)
  KAFKA-10186; Abort transaction with pending data with TransactionAbortedException (apache#9280)
  MINOR: Remove `TargetVoters` from `DescribeQuorum` (apache#9376)
  Revert "KAFKA-10469: Resolve logger levels hierarchically (apache#9266)"
  MINOR: Don't publish javadocs for raft module (apache#9336)
  KAFKA-9929: fix: add missing default implementations (apache#9321)
  KAFKA-10188: Prevent SinkTask::preCommit from being called after SinkTask::stop (apache#8910)
  KAFKA-10338; Support PEM format for SSL key and trust stores (KIP-651) (apache#9345)
  KAFKA-10527; Voters should not reinitialize as leader in same epoch (apache#9348)
  MINOR: Refactor unit tests around RocksDBConfigSetter (apache#9358)
  KAFKA-6733: Printing additional ConsumerRecord fields in DefaultMessageFormatter (apache#9099)
  MINOR: Annotate test BlockingConnectorTest as integration test (apache#9379)
  MINOR: Fix failing test due to KAFKA-10556 PR (apache#9372)
  KAFKA-10439: Connect's Values to parse BigInteger as Decimal with zero scale. (apache#9320)
  ...
mattwong949 pushed a commit to confluentinc/kafka that referenced this pull request Oct 13, 2020
* Updating trunk versions after cutting branch for 2.7

* KAFKA-9929: Support backward iterator on SessionStore (apache#9139)

Implements KIP-617 for `SessionStore`

Reviewers: A. Sophie Blee-Goldman <[email protected]>, John Roesler <[email protected]>

* MINOR: remove unused scala files from core module (apache#9296)


Reviewers: Mickael Maison <[email protected]>, Lee Dongjin <[email protected]>

* MINOR: correct package of LinuxIoMetricsCollector (apache#9271)


Reviewers: Mickael Maison <[email protected]>, Lee Dongjin <[email protected]>

* KAFKA-10028: Minor fixes to describeFeatures and updateFeatures apis (apache#9393)

In this PR, I have addressed the review comments from @chia7712 in apache#9001 which were provided after apache#9001 was merged. The changes are made mainly to KafkaAdminClient:

Improve error message in updateFeatures api when feature name is empty.
Propagate top-level error message in updateFeatures api.
Add an empty-parameter variety for describeFeatures api.
Minor documentation updates to @param and @return to make these resemble other apis.

Reviewers: Chia-Ping Tsai [email protected], Jun Rao [email protected]

* KAFKA-10271: Performance regression while fetching a key from a single partition (apache#9020)

StreamThreadStateStoreProvider excessive loop over calling internalTopologyBuilder.topicGroups(), which is synchronized, thus causing significant performance degradation to the caller, especially when store has many partitions.

Reviewers: John Roesler <[email protected]>, Guozhang Wang <[email protected]>

Co-authored-by: Jorge Esteban Quilcate Otoya <[email protected]>
Co-authored-by: Chia-Ping Tsai <[email protected]>
Co-authored-by: Kowshik Prakasam <[email protected]>
Co-authored-by: Dima Reznik <[email protected]>
Sign up for free to join this conversation on GitHub. Already have an account? Sign in to comment
Labels
core Kafka Broker
Projects
None yet
Development

Successfully merging this pull request may close these issues.

5 participants