Commit Graph

2782 Commits

Author SHA1 Message Date
Alyssa Huang 5b5f6fcafb
[KAFKA-15137] Do not log entire request payload in KRaftControllerChannelManager (#13988)
Reviewers: David Arthur <mumrah@gmail.com>
2023-07-11 10:48:53 +02:00
Cheryl Simmons e98508747a
Doc fixes: Fix format and other small errors in config documentation (#13661)
Various formatting fixes in the config docs.

Reviewers: Bill Bejeck <bbejeck@apache.org>
2023-07-10 12:48:35 -04:00
Lianet Magrans 4a61b48d3d
KAFKA-14966; [2/N] Extract OffsetFetcher reusable logic (#13898)
This is a follow up on the initial OffsetFetcher refactoring to extract reusable logic, needed for the new consumer implementation (initial refactoring merged with PR-13815).

Similar to the initial refactoring, this PR brings no changes to the existing logic, just extracting functions or pieces of logic.

There were no individual tests for the extracted functions, so no tests were migrated.

Reviewers: Jun Rao <junrao@gmail.com>
2023-07-05 17:20:49 -07:00
Gantigmaa Selenge b2d647904c
KAFKA-8982: Add retry of fetching metadata to Admin.deleteRecords (#13760)
Use AdminApiDriver class to refresh the metadata and retry the request that failed with retriable errors.

Reviewers: Luke Chen <showuon@gmail.com>, Divij Vaidya <diviv@amazon.com>, Mickael Maison <mmaison@redhat.com>, Dimitar Dimitrov <30328539+dimitarndimitrov@users.noreply.github.com>
2023-07-03 09:13:55 +08:00
Ismael Juma 1f4cbc5d53
MINOR: Add JDK 20 CI build and remove some branch builds (#12948)
It's good for us to add support for Java 20 in preparation for Java 21 - the next LTS.

Given that Scala 2.12 support has been deprecated, a Scala 2.12 variant is not included.

Also remove some branch builds that add load to the CI, but have
low value: JDK 8 & Scala 2.13 (JDK 8 support has been deprecated),
JDK 11 & Scala 2.12 (Scala 2.12 support has been deprecated) and
JDK 17 & Scala 2.12 (Scala 2.12 support has been deprecated).

A newer version of Mockito (4.9.0 -> 4.11.0) is required for Java 20 support, but we
only use it with Scala 2.13+ since it causes compilation errors with Scala 2.12. Similarly,
we upgrade easymock when the Java version is 16 or newer as it's incompatible
with powermock (which doesn't support Java 16 or newer).

Filed KAFKA-15117 for a test that fails with Java 20 (SslTransportLayerTest.testValidEndpointIdentificationCN).

Finally, fixed some lossy conversions that were added after #13582 was submitted.

Reviewers: Ismael Juma <ismael@juma.me.uk>
2023-06-30 01:12:00 -07:00
Kirk True a81f35c1c8
KAFKA-14831: Illegal state errors should be fatal in transactional producer (#13591)
Poison the transaction manager if we detect an illegal transition in the Sender thread. A ThreadLocal in is stored in TransactionManager so that the Sender can inform TransactionManager which thread it's using.

Reviewers: Daniel Urban <durban@cloudera.com>, Justine Olshan <jolshan@confluent.io>, Jason Gustafson <jason@confluent.io>
2023-06-29 11:21:15 -07:00
Bo Gao 005416879e
KAFKA-15053: Use case insensitive validator for security.protocol config (#13831)
Fixed a regression described in KAFKA-15053 that security.protocol only allows uppercase values like PLAINTEXT, SSL, SASL_PLAINTEXT, SASL_SSL. With this fix, both lower case and upper case values will be supported (e.g. PLAINTEXT, SSL, SASL_PLAINTEXT, SASL_SSL, plaintext, ssl, sasl_plaintext, sasl_ssl)

Reviewers: Chris Egerton <chrise@aiven.io>, Divij Vaidya <diviv@amazon.com>
2023-06-29 10:13:21 +02:00
José Armando García Sancio ee88a3d1b9
MINOR; Failed atomic file move should be logged at WARN (#13917)
When Kafka fails to perform an atomic file move the error is getting swallowed. Kafka should log these cases at least at WARN level.

Reviewers: Ron Dagostino <rndgstn@gmail.com>, Kirk True <kirk@kirktrue.pro>
2023-06-28 15:55:50 -07:00
Justine Olshan 2f71708955
KAFKA-15028: AddPartitionsToTxnManager metrics (#13798)
Adding the following metrics as per kip-890:

VerificationTimeMs – number of milliseconds from adding partition info to the manager to the time the response is sent. This will include the round trip to the transaction coordinator if it is called. This will also account for verifications that fail before the coordinator is called.

VerificationFailureRate – rate of verifications that returned in failure either from the AddPartitionsToTxn response or through errors in the manager.

AddPartitionsToTxnVerification metrics – separating the verification request metrics from the typical add partitions ones similar to how fetch replication and fetch consumer metrics are separated.

Reviewers: Divij Vaidya <diviv@amazon.com>
2023-06-28 09:00:37 -07:00
minjian.cai e71f68d6c9
MINOR: fix typos for client (#13884)
Reviewers: Luke Chen <showuon@gmail.com>, Divij Vaidya <diviv@amazon.com>, Kirk True <ktrue@confluent.io>
2023-06-28 16:47:42 +02:00
Manyanda Chitimbo c5889fcedd
MINOR: Split ConsumerCoordinator#testCommitOffsetMetadata onto two test cases testing commitSync and commitAsync (#13665)
Split ConsumerCoordinator#testCommitOffsetMetadata onto two test cases testing commitSync and commitAsync 

Reviewers:  Luke Chen <showuon@gmail.com>
2023-06-24 12:32:21 +08:00
Jeff Kim 1dbcb7da9e
KAFKA-14694: RPCProducerIdManager should not wait on new block (#13267)
RPCProducerIdManager initiates an async request to the controller to grab a block of producer IDs and then blocks waiting for a response from the controller.

This is done in the request handler threads while holding a global lock. This means that if many producers are requesting producer IDs and the controller is slow to respond, many threads can get stuck waiting for the lock.

This patch aims to:
* resolve the deadlock scenario mentioned above by not waiting for a new block and returning an error immediately
* remove synchronization usages in RpcProducerIdManager.generateProducerId()
* handle errors returned from generateProducerId() so that KafkaApis does not log unexpected errors
* confirm producers backoff before retrying
* introduce backoff if manager fails to process AllocateProducerIdsResponse

Reviewers: Artem Livshits <alivshits@confluent.io>, Jason Gustafson <jason@confluent.io>
2023-06-22 10:19:39 -07:00
Ismael Juma 9c8aaa2c35
MINOR: Fix lossy conversions flagged by Java 20 (#13582)
An example of the warning:
> warning: [lossy-conversions] implicit cast from long to int in compound assignment is possibly lossy

There should be no change in behavior as part of these changes - runtime logic ensured
we didn't run into issues due to the lossy conversions.

Reviewers: Divij Vaidya <diviv@amazon.com>
2023-06-22 08:05:55 -07:00
Mickael Maison 190e6de809
MINOR: Close code tag in Producer configs (#13875)
Reviewers: Divij Vaidya <diviv@amazon.com>
2023-06-19 16:58:55 +02:00
Daniel Scanteianu 6f7682d2f4
MINOR: add comment that timestamp unit is milliseconds (#13341)
Reviewers:  Divij Vaidya <diviv@amazon.com>
2023-06-19 13:22:09 +02:00
Luke Chen 74238656dc
KAFKA-15066: add "remote.log.metadata.manager.listener.name" config to rlmm (#13828)
add "remote.log.metadata.manager.listener.name" config to rlmm to allow producer/consumer to connect to the server. Also add tests.

Reviewers: Divij Vaidya <diviv@amazon.com>, Satish Duggana <satishd@apache.org>
2023-06-16 20:56:13 +08:00
Sushant Mahajan 5afce2de68
KAFKA-15077: Code to trim token in FileTokenRetriever (#13835)
The FileTokenRetriever class is used to read the access_token from a file on the clients system and then it is passed along with the jaas config to the OAuthBearerSaslServer. In case the token was sent using FileTokenRetriever on the client side, some EOL character is getting appended to the token, causing authentication to fail with the message:


Reviewers: Manikumar Reddy <manikumar.reddy@gmail.com>
2023-06-11 11:52:25 +05:30
Lianet Magrans 4af4bccbbf
KAFKA-14966: Extract OffsetFetcher reusable logic (#13815)
The OffsetFetcher is internally used by the KafkaConsumer to fetch offsets, validate and reset positions. For the new KafkaConsumer with a refactored threading model, similar functionality will be needed.

This is an initial refactoring for extracting logic from the OffsetFetcher, that will be reused by the new consumer implementation. No changes to the existing logic, just extracting classes, functions or pieces of logic.

All the functionality moved out of the OffsetFetcher is already covered by tests in OffsetFetcherTest and FetcherTest. There were no individual tests for the extracted functions, so no tests were migrated.

Reviewers: Jun Rao <junrao@gmail.com>
2023-06-08 14:03:45 -07:00
Lucas Brutschy ff77b3ad04
KAFKA-14278: Fix InvalidProducerEpochException and InvalidTxnStateException handling in producer clients (#13811)
This PR fixes three issues:

InvalidProducerEpochException was not handled consistently. InvalidProducerEpochException used to be able to be return via both transactional response and produce response, but as of KIP-588 (2.7+), transactional responses should not return InvalidProducerEpochException anymore, only produce responses can. It can happen that older brokers may still return InvalidProducerEpochException for transactional responses; these must be converted to the newer ProducerFencedException. This conversion wasn't done for TxnOffsetCommit (sent to the group coordinator).

InvalidTxnStateException was double-wrapped in KafkaException, whereas other exceptions are usually wrapped only once. Furthermore, InvalidTxnStateException was not handled at all for in AddOffsetsToTxn response, where it should be a possible error as well, according to API documentation.

According to API documentation, UNSUPPORTED_FOR_MESSAGE_FORMAT is not possible for TxnOffsetCommit, but it looks like it is, and it is being handled there, so I updated the API documentation.

Reviewers: Justine Olshan <jolshan@confluent.io>
2023-06-07 09:48:14 -07:00
Erik van Oosten 59d30a06fc
KAFKA-10337: await async commits in commitSync even if no offsets given (#13678)
The contract for Consumer#commitSync() guarantees that the callbacks for all prior async commits will be invoked before it returns. Prior to this patch the contract could be violated if an empty offsets map were passed in to Consumer#commitSync().

Reviewers: Philip Nee <philipnee@gmail.com>, David Jacot <djacot@confluent.io>
2023-06-07 16:55:03 +02:00
David Jacot 7d147cf241
KAFKA-14462; [14/N] Add PartitionWriter (#13675)
This patch introduces the `PartitionWriter` interface in the `group-coordinator` module. The `ReplicaManager` resides in the `core` module and it is thus not accessible from the `group-coordinator` one. The `CoordinatorPartitionWriter` is basically an implementation of the interface residing in `core` which interfaces with the `ReplicaManager`.

One notable difference from the usual produce path is that the `PartitionWriter` returns the offset following the written records. This is then used by the coordinator runtime to track when the request associated with the write can be completed.

Reviewers: Jeff Kim <jeff.kim@confluent.io>, Justine Olshan <jolshan@confluent.io>
2023-06-06 16:24:48 +02:00
mojh7 04f2f6a26a
MINOR: Typo and unused method removal (#13739)
clean up unused private method and removed typos

Reviewers:  Divij Vaidya <diviv@amazon.com>,  Manyanda Chitimbo <manyanda.chitimbo@gmail.com>,  Daniel Scanteianu, Josep Prat <josep.prat@aiven.io>
2023-06-06 10:50:56 +02:00
Gabriel Oliveira 443bd1dd82
MINOR: Add "versions" tag to recently added ReplicaState field on Fetch Request (#13680)
Reviewers: David Jacot <djacot@confluent.io>
2023-06-05 13:40:20 +02:00
Divij Vaidya fe6a827e20
KAFKA-14633: Reduce data copy & buffer allocation during decompression (#13135)
After this change,

    For broker side decompression: JMH benchmark RecordBatchIterationBenchmark demonstrates 20-70% improvement in throughput (see results for RecordBatchIterationBenchmark.measureSkipIteratorForVariableBatchSize).
    For consumer side decompression: JMH benchmark RecordBatchIterationBenchmark a mix bag of single digit regression for some compression type to 10-50% improvement for Zstd (see results for RecordBatchIterationBenchmark.measureStreamingIteratorForVariableBatchSize).

Reviewers: Luke Chen <showuon@gmail.com>, Manyanda Chitimbo <manyanda.chitimbo@gmail.com>, Ismael Juma <mail@ismaeljuma.com>
2023-06-05 15:04:49 +08:00
Daniel Scanteianu 6d72c26731
KAFKA-14841 Handle callbacks to ConsumerRebalanceListener in MockConsumer (#13455)
Reviewers: Philip Nee <philipnee@gmail.com>
2023-05-26 08:33:03 -05:00
Manyanda Chitimbo c14e0df617
KAFKA-14961: harden DefaultBackgroundThreadTest.testStartupAndTearDown test (#13664)
1. Ensures that NPE are not thrown
2. Ensures that the background thread has been started to avoid flasky
   assertions failures on isRunning
3. Add a check that the thread is not running when closed

Reviewers: Philip Nee <philipnee@gmail.com>, Kirk True <kirk@kirktrue.pro>
2023-05-26 08:28:03 -05:00
Aaron Ai c90a08c37e
MINOR: Fix the outdated comments of ConfigDef (#13710)
Fix the outdated comments of ConfigDef since the signature of the corresponding method has been updated.

Reviewers: Luke Chen <showuon@gmail.com>
2023-05-19 13:00:20 +08:00
Divij Vaidya bb10ae4273
KAFKA-14962: Trim whitespace from ACL configuration (#13670)
Reviewers: Manikumar Reddy <manikumar.reddy@gmail.com>, Christo Lolov <lolovc@amazon.com>
2023-05-12 23:51:00 +05:30
dengziming a7c9842f70
KAFKA-14291: KRaft controller should return right finalized features in ApiVersionResponse (#13679)
The KRaft controller return empty finalized features in `ApiVersionResponse`, the brokers are not infected by this, so this problem doesn't have any impact currently, but it's worth fixing it to avoid unexpected problems.

And there is a bunch of of confusing methods in `ApiVersionResponse` which are only used in test code, I moved them to TestUtils to make the code more clear, and force everyone to pass in the correct parameters instead of the default zero parameters, for example, empty supported features and empty finalized features.

Reviewers: Luke Chen <showuon@gmail.com>
2023-05-12 13:46:06 +08:00
vamossagar12 86daf8ce65
KAFKA-14913: Using ThreadUtils.shutdownExecutorServiceQuietly to close executors in Connect Runtime (#13594)
#13557 introduced a utils method to close executors silently. This PR leverages that method to close executors in connect runtime. There was duplicate code while closing the executors which isn't the case with this PR.

Note that there are a few more executors used in Connect runtime but their close methods don't follow this pattern of shutdown, await and shutdown. Some of them have some logic like executor like Worker, so not changing at such places.

---------

Co-authored-by: Sagar Rao <sagarrao@Sagars-MacBook-Pro.local>

Reviewers: Daniel Urban <durban@cloudera.com>, Yash Mayya <yash.mayya@gmail.com>, Viktor Somogyi-Vass <viktorsomogyi@gmail.com>
2023-05-08 16:39:47 +02:00
Federico Valeri 2607e0edb7
MINOR: Fix producer Callback comment (#13669)
Fixes the wrong exception name: OffsetMetadataTooLargeException.

Reviewers: Manyanda Chitimbo <manyanda.chitimbo@gmail.com>, Luke Chen <showuon@gmail.com>
2023-05-08 14:33:31 +08:00
Chia-Ping Tsai 6e7144ac24
MINOR: add docs to remind reader that impl of ConsumerPartitionAssign… (#13659)
Reviewers: David Jacot <djacot@confluent.io>, Kirk True <kirk@kirktrue.pro>
2023-05-06 02:56:26 +08:00
Divij Vaidya 6bcc497c36
KAFKA-14766: Improve performance of VarInt encoding and decoding (#13312)
Motivation

Reading/writing the protocol buffer varInt32 and varInt64 (also called varLong in our code base) is in the hot path of data plane code in Apache Kafka. We read multiple varInt in a record and in long. Hence, even a minor change in performance could extrapolate to larger performance benefit.

In this PR, we only update varInt32 encoding/decoding.
Changes

This change uses loop unrolling and reduces the amount of repetition of calculations. Based on the empirical results from the benchmark, the code has been modified to pick up the best implementation.
Results

Performance has been evaluated using JMH benchmarks on JDK 17.0.6. Various implementations have been added in the benchmark and benchmarking has been done for different sizes of varints and varlongs. The benchmark for various implementations have been added at ByteUtilsBenchmark.java

Reviewers: Ismael Juma <mlists@juma.me.uk>, Luke Chen <showuon@gmail.com>, Alexandre Dupriez <alexandre.dupriez@gmail.com>
2023-05-05 20:05:20 +08:00
Justine Olshan ffd814d25f
KAFKA-14916: Fix code that assumes transactional ID implies all records are transactional (#13607)
Also modifies verification to only add a partition to verify if it is transactional.

When verifying we look at all the transactional producer IDs and throw INVALID_RECORD on the request if one is different.

Reviewers: Kirk True <ktrue@confluent.io>, Artem Livshits <alivshits@confluent.io>, Jason Gustafson <jason@confluent.io>
2023-05-04 09:55:45 -07:00
Philip Nee ea81e99e59
KAFKA-13668: Retry upon missing initProducerId due to authorization error (#12149)
Producers used to throw a fatal error upon failing initProducerId, which can be caused by authorization errors. In this case, the user will need to instantiate a producer.

This PR makes authorization errors non-fatal so that the user can retry until the permission is fixed by an admin.

Here we first transition the producer to the ABORTABLE state, then to the UNINITIALIZED state (so that the producer is recoverable). Upon the subsequent send, the producer will transition to INITIALIZING and attempt to send another InitProducerIdRequest.

Reviewers: Kirk True <ktrue@confluent.io>, David Jacot <djacot@confluent.io>, Jason Gustafson <jason@confluent.io>, Justine Olshan <jolshan@confluent.io>
2023-05-04 09:20:01 -07:00
Christo Lolov dc7819d7f1
KAFKA-14594: Move LogDirsCommand to tools module (#13122)
Reviewers: Mickael Maison <mickael.maison@gmail.com>
2023-05-04 12:00:33 +02:00
David Mao d46c3f259c
MINOR: Reduce number of threads created for integration test brokers (#13655)
The integration tests seem to create an unnecessarily large number of threads. This reduces the number of threads created per integration test harness broker.

Reviewers: Luke Chen <showuon@gmail.com>. Justine Olshan <jolshan@confluent.io>
2023-05-03 17:09:43 -07:00
Jason Gustafson c08120f83f
MINOR: Allow tagged fields with version subset of flexible version range (#13551)
The generated message types are missing a range check for the case when the tagged version range is a subset of
the flexible version range. This causes the tagged field count, which is computed correctly, to conflict with the
number of tags serialized.

Reviewers: Colin P. McCabe <cmccabe@apache.org>
2023-05-03 15:25:32 -07:00
Christo Lolov f44ee4fab7
MINOR: Remove unnecessary code in client/connect (#13259)
Reviewers: Mickael Maison <mickael.maison@gmail.com>
2023-05-02 17:39:31 +02:00
Philip Nee 64ebbc577d
MINOR: Fixing typos in the ConsumerCoordinator (#13618)
Reviewers: Divij Vaidya <diviv@amazon.com>, Christo Lolov <lolovc@amazon.com>, David Jacot <djacot@confluent.io>
2023-04-28 17:46:40 +02:00
Anton Agestam e55fbceb66
MINOR: Fix incorrect description of SessionLifetimeMs (#13649)
Reviewers: Mickael Maison <mickael.maison@gmail.com>
2023-04-28 12:54:28 +02:00
Philip Nee c6ad151ac3
KAFKA-14639: A single partition may be revoked and assign during a single round of rebalance (#13550)
This is a really long story, but the incident started in KAFKA-13419 when we observed a member sending out a topic partition owned from the previous generation when a member missed a rebalance cycle due to REBALANCE_IN_PROGRESS.

This patch changes the AbstractStickyAssignor.AllSubscriptionsEqual method.  In short, it should no long check and validate only the highest generation.  Instead, we consider 3 cases:
1. Member will continue to hold on to its partition if there are no other owners
2. If there are 1+ owners to the same partition. One with the highest generation will win.
3. If two members of the same generation hold on to the same partition.  We will log an error but remove both from the assignment. (Same with the current logic)

Here are some important notes that lead to the patch:
- If a member is kicked out of the group, and `UNKNOWN_MEMBER_ID` will be thrown.
- It seems to be a common situation that members are late to joinGroup and therefore get `REBALANCE_IN_PROGRESS` error.  This is why we don't want to reset generation because it might cause lots of revocations and can be disruptive

To summarize the current behavior of different errors:
`REBALANCE_IN_PROGRESS`
- heartbeat: requestRejoin if member state is stable
- joinGroup: rejoin immediately
- syncGroup: rejoin immediately
- commit: requestRejoin and fail the commit. Raise this exception if the generation is staled, i.e. another rebalance is already in progress.

`UNKNOWN_MEMBER_ID`
- heartbeat: resetStateAndRejoinif generation hasn't changed. otherwise, ignore
- joinGroup: resetStateAndRejoin if generation unchanged, otherwise rejoin immediately
- syncGroup:  resetStateAndRejoin if generation unchanged, otherwise rejoin immediately

`ILLEGAL_GENERATION`
- heartbeat: resetStateAndRejoinif generation hasn't changed. otherwise, ignore
- syncGroup: raised the exception if generation has been resetted or the member hasn't completed rebalancing.  then resetStateAndRejoin if generation unchanged, otherwise rejoin immediately

Reviewers: David Jacot <djacot@confluent.io>
2023-04-28 11:08:32 +02:00
Luke Chen d796480fe8
KAFKA-14909: check zkMigrationReady tag before migration (#13631)
1. add ZkMigrationReady in apiVersionsResponse
2. check all nodes if ZkMigrationReady are ready before moving to next migration state

Reviewers: David Arthur <mumrah@gmail.com>, dengziming <dengziming1993@gmail.com>
2023-04-28 14:35:12 +08:00
Colin P. McCabe 7049333617 KAFKA-14943: Fix ClientQuotaControlManager validation
Don't allow setting negative or zero values for quotas. Don't allow SCRAM mechanism names to be
used as client quota names. SCRAM mechanisms are not client quotas. (The confusion arose because of
internal ZK representation details that treated them both as "client configs.")

Add unit tests for ClientQuotaControlManager.isValidIpEntity and
ClientQuotaControlManager.configKeysForEntityType.

This change doesn't affect metadata record application, only input validation. If there are bad
client quotas that are set currently, this change will not alter the current behavior (of throwing
an exception and ignoring the bad quota).
2023-04-27 10:42:32 -07:00
LinShunKang dd6690a7a0
KAFKA-14944: Reduce CompletedFetch#parseRecord() memory copy (#12545)
This implements KIP-863: https://cwiki.apache.org/confluence/pages/viewpage.action?pageId=225152035
Direct use ByteBuffer instead of byte[] to deserialize.

Reviewers: Luke Chen <showuon@gmail.com>, Kirk True <kirk@kirktrue.pro>
2023-04-27 10:44:08 +08:00
Manyanda Chitimbo d83a734c41
MINOR: only set sslEngine#setUseClientMode to false once when ssl mode is server (#13626)
The sslEngine.setUseClientMode(false) was duplicated when ssl mode is server during SSLEngine creation
in DefaultSslEngineFactory.java. The patch attemps to remove the duplicated call.

Reviewers:   maulin-vasavada <maulin.vasavada@gmail.com>, Divij Vaidya <diviv@amazon.com>, Manikumar Reddy <manikumar.reddy@gmail.com>
2023-04-25 15:32:24 +05:30
Dimitar Dimitrov e14dd8024a
KAFKA-14821 Implement the listOffsets API with AdminApiDriver (#13432)
We are handling complex workflows ListOffsets by chaining together MetadataCall instances and ListOffsetsCall instances, there are many complex and error-prone logic. In this PR we rewrote it with the `AdminApiDriver` infra, notable changes better than old logic:
1. Retry lookup stage on receiving `NOT_LEADER_OR_FOLLOWER` and `LEADER_NOT_AVAILABLE`, whereas in the past we failed the partition directly without retry.
2. Removing class field `supportsMaxTimestamp` and calculating it on the fly to avoid the mutable state, this won't change any behavior of  the client.
3. Retry fulfillment stage on `RetriableException`, whereas in the past we just retry fulfillment stage on `InvalidMetadataException`, this means we will retry on `TimeoutException` and other `RetriableException`.

We also `handleUnsupportedVersionException` to `AdminApiHandler` and `AdminApiLookupStrategy`, they are used to keep consistency with old logic, and we can continue improvise them. 

Reviewers: Ziming Deng <dengziming1993@gmail.com>, David Jacot <djacot@confluent.io>
2023-04-20 11:29:27 +08:00
Dániel Urbán 454b72161a
KAFKA-14902: KafkaStatusBackingStore retries on a dedicated background thread to avoid stack overflows (#13557)
KafkaStatusBackingStore uses an infinite retry logic on producer send, which can lead to a stack overflow.
To avoid the problem, a background thread was added, and the callback submits the retry onto the background thread.
2023-04-18 09:40:14 +02:00
Justine Olshan 56dcb837a2
KAFKA-14561: Improve transactions experience for older clients by ensuring ongoing transaction (#13391)
Added check for ongoing transaction
Thread to send and receive verify only add partition to txn requests
Code to send on request thread courtesy of @artemlivshits

Reviewers: Artem Livshits <alivshits@confluent.io>, Jun Rao <junrao@gmail.com>
2023-04-12 17:04:51 -07:00
Rajini Sivaram b64ac94a8c
KAFKA-14891: Fix rack-aware range assignor to assign co-partitioned subsets (#13539)
Reviewers: David Jacot <djacot@confluent.io>
2023-04-12 08:35:03 +01:00