Re: [PR] KAFKA-16608: Honour interrupted thread state on KafkaConsumer.poll [kafka]

2024-05-01 Thread via GitHub
AndrewJSchofield commented on code in PR #15803: URL: https://github.com/apache/kafka/pull/15803#discussion_r1587067096 ## clients/src/main/java/org/apache/kafka/clients/consumer/internals/FetchBuffer.java: ## @@ -173,15 +173,23 @@ void awaitNotEmpty(Timer timer) {

Re: [PR] KAFKA-16511: Fix the leaking tiered segments during segment deletion [kafka]

2024-05-01 Thread via GitHub
showuon commented on code in PR #15817: URL: https://github.com/apache/kafka/pull/15817#discussion_r1587041449 ## core/src/main/java/kafka/log/remote/RemoteLogManager.java: ## @@ -1217,26 +1217,41 @@ public String toString() { * @return true if the remote segment's

Re: [PR] KAFKA-16573: Specify node and store where serdes are needed [kafka]

2024-05-01 Thread via GitHub
mjsax commented on code in PR #15790: URL: https://github.com/apache/kafka/pull/15790#discussion_r1587028687 ## streams/src/main/java/org/apache/kafka/streams/processor/internals/SinkNode.java: ## @@ -58,8 +60,14 @@ public void addChild(final ProcessorNode child) { public

Re: [PR] KAFKA-13152: Kip 770 buffer size fix [kafka]

2024-05-01 Thread via GitHub
mjsax commented on PR #13283: URL: https://github.com/apache/kafka/pull/13283#issuecomment-2089475025 Very happy to see some activity on this PR. The release plan is in the wiki: https://cwiki.apache.org/confluence/display/KAFKA/Release+Plan+3.8.0 Feature freeze is May/29, so there

Re: [PR] DO NOT MERGE: Isolate Connect tests [kafka]

2024-05-01 Thread via GitHub
github-actions[bot] commented on PR #15229: URL: https://github.com/apache/kafka/pull/15229#issuecomment-2089469362 This PR is being marked as stale since it has not had any activity in 90 days. If you would like to keep this PR alive, please ask a committer for review. If the PR has

Re: [PR] KAFKA-15951: MissingSourceTopicException should include topic names [kafka]

2024-05-01 Thread via GitHub
mjsax commented on code in PR #15573: URL: https://github.com/apache/kafka/pull/15573#discussion_r1587018485 ## streams/src/main/java/org/apache/kafka/streams/processor/internals/assignment/AssignorError.java: ## @@ -19,20 +19,31 @@ public enum AssignorError { // Note:

[jira] [Updated] (KAFKA-16260) Deprecate window.size.ms and window.inner.serde.class in StreamsConfig

2024-05-01 Thread Matthias J. Sax (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-16260?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Matthias J. Sax updated KAFKA-16260: Component/s: clients > Deprecate window.size.ms and window.inner.serde.class in

[jira] [Updated] (KAFKA-16260) Deprecate window.size.ms and window.inner.serde.class in StreamsConfig

2024-05-01 Thread Matthias J. Sax (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-16260?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Matthias J. Sax updated KAFKA-16260: Description: {{window.size.ms}} and {{window.inner.serde.class}} are not a true

[jira] [Updated] (KAFKA-16260) Deprecate window.size.ms and window.inner.serde.class in StreamsConfig

2024-05-01 Thread Matthias J. Sax (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-16260?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Matthias J. Sax updated KAFKA-16260: Description: {{indwindow.size.ms}} and `is not a true KafkaStreams config, and results in

[jira] [Updated] (KAFKA-16260) Deprecate window.size.ms and inner.serde.class in StreamsConfig

2024-05-01 Thread Matthias J. Sax (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-16260?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Matthias J. Sax updated KAFKA-16260: Description: {{window.size.ms}} and `is not a true KafkaStreams config, and results in an

[jira] [Updated] (KAFKA-16260) Deprecate window.size.ms and window.inner.serde.class in StreamsConfig

2024-05-01 Thread Matthias J. Sax (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-16260?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Matthias J. Sax updated KAFKA-16260: Summary: Deprecate window.size.ms and window.inner.serde.class in StreamsConfig (was:

[jira] [Updated] (KAFKA-16260) Deprecate window.size.ms and inner.serde.class in StreamsConfig

2024-05-01 Thread Matthias J. Sax (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-16260?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Matthias J. Sax updated KAFKA-16260: Summary: Deprecate window.size.ms and inner.serde.class in StreamsConfig (was: Deprecate

[jira] [Updated] (KAFKA-16260) Deprecate window.size.ms in StreamsConfig

2024-05-01 Thread Matthias J. Sax (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-16260?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Matthias J. Sax updated KAFKA-16260: Description: {{window.size.ms}}  is not a true KafkaStreams config, and results in an

[jira] [Updated] (KAFKA-16260) Deprecate window.size.ms in StreamsConfig

2024-05-01 Thread Matthias J. Sax (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-16260?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Matthias J. Sax updated KAFKA-16260: Labels: KIP (was: needs-kip) > Deprecate window.size.ms in StreamsConfig >

Re: [PR] KAFKA-16572: allow defining number of disks per broker in ClusterTest [kafka]

2024-05-01 Thread via GitHub
chia7712 commented on code in PR #15745: URL: https://github.com/apache/kafka/pull/15745#discussion_r1587006765 ## core/src/test/java/kafka/test/ClusterConfig.java: ## @@ -55,14 +56,21 @@ public class ClusterConfig { private final Map> perBrokerOverrideProperties;

[jira] [Commented] (KAFKA-15561) Client support for new SubscriptionPattern based subscription

2024-05-01 Thread Phuc Hong Tran (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-15561?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=17842828#comment-17842828 ] Phuc Hong Tran commented on KAFKA-15561: [~kirktrue] just a quick question, why was this moved

Re: [PR] KAFKA-16427: KafkaConsumer#position() does not respect timeout when group protocol is CONSUMER [kafka]

2024-05-01 Thread via GitHub
chia7712 commented on code in PR #15843: URL: https://github.com/apache/kafka/pull/15843#discussion_r1587003306 ## clients/src/main/java/org/apache/kafka/clients/consumer/internals/AsyncKafkaConsumer.java: ## @@ -906,6 +906,7 @@ public long position(TopicPartition partition,

Re: [PR] KAFKA-16427: KafkaConsumer#position() does not respect timeout when group protocol is CONSUMER [kafka]

2024-05-01 Thread via GitHub
chia7712 commented on code in PR #15843: URL: https://github.com/apache/kafka/pull/15843#discussion_r1586996403 ## clients/src/main/java/org/apache/kafka/clients/consumer/internals/AsyncKafkaConsumer.java: ## @@ -906,6 +906,7 @@ public long position(TopicPartition partition,

[jira] [Commented] (KAFKA-16655) deflake ZKMigrationIntegrationTest.testDualWrite

2024-05-01 Thread Alyssa Huang (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-16655?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=17842820#comment-17842820 ] Alyssa Huang commented on KAFKA-16655: -- [https://github.com/apache/kafka/pull/15845/files]  >

[PR] KAFKA-16655 Deflaking ZKMigrationIntegrationTest.testDualWrite [kafka]

2024-05-01 Thread via GitHub
ahuang98 opened a new pull request, #15845: URL: https://github.com/apache/kafka/pull/15845 This test occasionally fails due to stale broker epoch exceptions, which in turn causes allocate producer ids to fail. Adds retries as stale broker epoch is a retriable issue, and fixes

[jira] [Created] (KAFKA-16655) deflake ZKMigrationIntegrationTest.testDualWrite

2024-05-01 Thread Alyssa Huang (Jira)
Alyssa Huang created KAFKA-16655: Summary: deflake ZKMigrationIntegrationTest.testDualWrite Key: KAFKA-16655 URL: https://issues.apache.org/jira/browse/KAFKA-16655 Project: Kafka Issue Type:

Re: [PR] KAFKA-16608: Honour interrupted thread state on KafkaConsumer.poll [kafka]

2024-05-01 Thread via GitHub
chia7712 commented on code in PR #15803: URL: https://github.com/apache/kafka/pull/15803#discussion_r1586968300 ## clients/src/main/java/org/apache/kafka/clients/consumer/internals/FetchBuffer.java: ## @@ -173,15 +173,23 @@ void awaitNotEmpty(Timer timer) { //

[PR] KAFKA-16637: KIP-848 does not work well [kafka]

2024-05-01 Thread via GitHub
kirktrue opened a new pull request, #15844: URL: https://github.com/apache/kafka/pull/15844 This issue is related to an optimization for offset fetch logic. When a user calls `Consumer.poll()`, among other things, the consumer performs a network request to fetch any

[jira] [Updated] (KAFKA-16637) KIP-848 does not work well

2024-05-01 Thread Kirk True (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-16637?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Kirk True updated KAFKA-16637: -- Priority: Blocker (was: Minor) > KIP-848 does not work well > -- > >

[jira] [Commented] (KAFKA-16427) KafkaConsumer#position() does not respect timeout when group protocol is CONSUMER

2024-05-01 Thread Kirk True (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-16427?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=17842817#comment-17842817 ] Kirk True commented on KAFKA-16427: --- This bug can be triggered with an embarrassingly simple

Re: [PR] MINOR: Replaced Utils.join() with JDK API. [kafka]

2024-05-01 Thread via GitHub
chia7712 commented on code in PR #15823: URL: https://github.com/apache/kafka/pull/15823#discussion_r1586962262 ## connect/runtime/src/main/java/org/apache/kafka/connect/util/TopicAdmin.java: ## @@ -399,7 +398,7 @@ public TopicCreationResponse createOrFindTopics(NewTopic...

[jira] [Commented] (KAFKA-16541) Potential leader epoch checkpoint file corruption on OS crash

2024-05-01 Thread Haruki Okada (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-16541?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=17842816#comment-17842816 ] Haruki Okada commented on KAFKA-16541: -- [~junrao] Yes. My concern now is only changing renameDir

[PR] KAFKA-16427: KafkaConsumer#position() does not respect timeout when group protocol is CONSUMER [kafka]

2024-05-01 Thread via GitHub
kirktrue opened a new pull request, #15843: URL: https://github.com/apache/kafka/pull/15843 The AsyncKafkaConsumer implementation of `position(TopicPartition, Duration)` was not updating its internal `Timer`, causing it to execute the loop forever. Adding a call to update the `Timer` at

[jira] [Resolved] (KAFKA-16647) Remove setMetadataDirectory from BrokerNode/ControllerNode

2024-05-01 Thread Chia-Ping Tsai (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-16647?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Chia-Ping Tsai resolved KAFKA-16647. Fix Version/s: 3.8.0 Resolution: Fixed > Remove setMetadataDirectory from

Re: [PR] KAFKA-16647: Remove setMetadataDirectory from BrokerNode/ControllerNode [kafka]

2024-05-01 Thread via GitHub
chia7712 merged PR #15833: URL: https://github.com/apache/kafka/pull/15833 -- This is an automated message from the Apache Git Service. To respond to the message, please log on to GitHub and use the URL above to go to the specific comment. To unsubscribe, e-mail:

Re: [PR] KAFKA-16647: Remove setMetadataDirectory from BrokerNode/ControllerNode [kafka]

2024-05-01 Thread via GitHub
chia7712 commented on PR #15833: URL: https://github.com/apache/kafka/pull/15833#issuecomment-2089361742 loop the failed tests on my local, and they pass. ``` ./gradlew cleanTest :connect:runtime:test --tests

Re: [PR] KAFKA-16223 Replace EasyMock/PowerMock with Mockito for KafkaConfigBackingStoreTest (2/3) [kafka]

2024-05-01 Thread via GitHub
chia7712 commented on code in PR #15841: URL: https://github.com/apache/kafka/pull/15841#discussion_r1586954293 ## connect/runtime/src/test/java/org/apache/kafka/connect/storage/KafkaConfigBackingStoreMockitoTest.java: ## @@ -347,6 +380,165 @@ public void

[jira] [Assigned] (KAFKA-16654) Refactor kafka.test.annotation.Type and ClusterTestExtensions

2024-05-01 Thread Chia-Ping Tsai (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-16654?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Chia-Ping Tsai reassigned KAFKA-16654: -- Assignee: TaiJuWu (was: Chia-Ping Tsai) > Refactor kafka.test.annotation.Type and

Re: [PR] KAFKA-15307: update/note deprecated configs [kafka]

2024-05-01 Thread via GitHub
mjsax commented on code in PR #14360: URL: https://github.com/apache/kafka/pull/14360#discussion_r1586935977 ## docs/streams/developer-guide/config-streams.html: ## @@ -240,24 +240,29 @@ num.standby.replicas - acceptable.recovery.lag + acceptable.recovery.lag

[jira] [Commented] (KAFKA-16654) Refactor kafka.test.annotation.Type and ClusterTestExtensions

2024-05-01 Thread TaiJuWu (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-16654?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=17842808#comment-17842808 ] TaiJuWu commented on KAFKA-16654: - Hi  [~chia7712] , If you are not working on it, could you assign it

[jira] [Commented] (KAFKA-16514) Kafka Streams: stream.close(CloseOptions) does not respect options.leaveGroup flag.

2024-05-01 Thread Matthias J. Sax (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-16514?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=17842805#comment-17842805 ] Matthias J. Sax commented on KAFKA-16514: - Thanks for the background! Makes sense. > Kafka

[jira] [Resolved] (KAFKA-16614) Disallow `@ClusterTemplate("")`

2024-05-01 Thread Chia-Ping Tsai (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-16614?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Chia-Ping Tsai resolved KAFKA-16614. Fix Version/s: 3.8.0 Resolution: Fixed > Disallow `@ClusterTemplate("")` >

Re: [PR] KAFKA-16614:Disallow @ClusterTemplate("") [kafka]

2024-05-01 Thread via GitHub
chia7712 merged PR #15800: URL: https://github.com/apache/kafka/pull/15800 -- This is an automated message from the Apache Git Service. To respond to the message, please log on to GitHub and use the URL above to go to the specific comment. To unsubscribe, e-mail:

[jira] [Created] (KAFKA-16654) Refactor kafka.test.annotation.Type and ClusterTestExtensions

2024-05-01 Thread Chia-Ping Tsai (Jira)
Chia-Ping Tsai created KAFKA-16654: -- Summary: Refactor kafka.test.annotation.Type and ClusterTestExtensions Key: KAFKA-16654 URL: https://issues.apache.org/jira/browse/KAFKA-16654 Project: Kafka

Re: [PR] KAFKA-16207; KRaft's internal log listener to update voter set [kafka]

2024-05-01 Thread via GitHub
jsancio commented on code in PR #15671: URL: https://github.com/apache/kafka/pull/15671#discussion_r1586871366 ## raft/src/main/java/org/apache/kafka/raft/KafkaRaftClient.java: ## @@ -2415,17 +2476,30 @@ public void resign(int epoch) { @Override public Optional>

[jira] [Updated] (KAFKA-14620) Add a type for SnapshotId

2024-05-01 Thread Jira
[ https://issues.apache.org/jira/browse/KAFKA-14620?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] José Armando García Sancio updated KAFKA-14620: --- Description: We have seen issues where the state machine assumes

Re: [PR] KAFKA-16207; KRaft's internal log listener to update voter set [kafka]

2024-05-01 Thread via GitHub
jsancio commented on code in PR #15671: URL: https://github.com/apache/kafka/pull/15671#discussion_r1586869606 ## raft/src/main/java/org/apache/kafka/raft/KafkaRaftClient.java: ## @@ -370,8 +363,52 @@ private void maybeFireLeaderChange() { } } -@Override -

Re: [PR] KAFKA-16207; KRaft's internal log listener to update voter set [kafka]

2024-05-01 Thread via GitHub
jsancio commented on code in PR #15671: URL: https://github.com/apache/kafka/pull/15671#discussion_r1586867970 ## raft/src/main/java/org/apache/kafka/raft/KafkaRaftClient.java: ## @@ -370,8 +363,52 @@ private void maybeFireLeaderChange() { } } -@Override -

[jira] [Created] (KAFKA-16653) Remove delayed initialization because of static voter set

2024-05-01 Thread Jira
José Armando García Sancio created KAFKA-16653: -- Summary: Remove delayed initialization because of static voter set Key: KAFKA-16653 URL: https://issues.apache.org/jira/browse/KAFKA-16653

Re: [PR] KAFKA-16207; KRaft's internal log listener to update voter set [kafka]

2024-05-01 Thread via GitHub
jsancio commented on code in PR #15671: URL: https://github.com/apache/kafka/pull/15671#discussion_r1586861717 ## raft/src/main/java/org/apache/kafka/raft/KafkaRaftClient.java: ## @@ -159,67 +165,76 @@ public class KafkaRaftClient implements RaftClient { private final

Re: [PR] KAFKA-16624: Don't generate useless PartitionChangeRecord on older MV [kafka]

2024-05-01 Thread via GitHub
cmccabe commented on code in PR #15810: URL: https://github.com/apache/kafka/pull/15810#discussion_r1586861400 ## metadata/src/main/java/org/apache/kafka/controller/PartitionChangeBuilder.java: ## @@ -365,45 +369,62 @@ private void tryElection(PartitionChangeRecord record) {

Re: [PR] KAFKA-16624: Don't generate useless PartitionChangeRecord on older MV [kafka]

2024-05-01 Thread via GitHub
cmccabe commented on PR #15810: URL: https://github.com/apache/kafka/pull/15810#issuecomment-2089185449 > Should we have a new test for that specific case? Good point. Added. -- This is an automated message from the Apache Git Service. To respond to the message, please log on to

Re: [PR] KAFKA-16207; KRaft's internal log listener to update voter set [kafka]

2024-05-01 Thread via GitHub
jsancio commented on code in PR #15671: URL: https://github.com/apache/kafka/pull/15671#discussion_r1586860365 ## raft/src/main/java/org/apache/kafka/raft/internals/BatchAccumulator.java: ## @@ -213,20 +213,21 @@ private void completeCurrentBatch() { * * @param

Re: [PR] KAFKA-16480: Bump ListOffsets version, IBP version and mark last version of ListOffsets as unstable [kafka]

2024-05-01 Thread via GitHub
junrao commented on PR #15673: URL: https://github.com/apache/kafka/pull/15673#issuecomment-2089169862 @clolov: Are you able to address the remaining comments? 3.8.0 code freeze is getting close. Thanks. -- This is an automated message from the Apache Git Service. To respond to the

Re: [PR] KAFKA-16649: Remove lock from DynamicBrokerConfig.removeReconfigurable [kafka]

2024-05-01 Thread via GitHub
cmccabe commented on PR #15838: URL: https://github.com/apache/kafka/pull/15838#issuecomment-2089167707 > Thanks for the patch, @cmccabe. I looked for other usages of this lock and see we're obtaining the write lock in DynamicBrokerConfig#updateBrokerConfig which gets called from

Re: [PR] KAFKA-16649: Remove lock from DynamicBrokerConfig.removeReconfigurable [kafka]

2024-05-01 Thread via GitHub
cmccabe commented on code in PR #15838: URL: https://github.com/apache/kafka/pull/15838#discussion_r1586844632 ## core/src/test/scala/integration/kafka/server/KRaftClusterTest.scala: ## @@ -1541,6 +1541,36 @@ class KRaftClusterTest { cluster.close() } } + +

Re: [PR] KAFKA-16649: Remove lock from DynamicBrokerConfig.removeReconfigurable [kafka]

2024-05-01 Thread via GitHub
cmccabe commented on code in PR #15838: URL: https://github.com/apache/kafka/pull/15838#discussion_r1586843624 ## core/src/main/scala/kafka/server/DynamicBrokerConfig.scala: ## @@ -303,17 +303,17 @@ class DynamicBrokerConfig(private val kafkaConfig: KafkaConfig) extends

[jira] [Commented] (KAFKA-16541) Potential leader epoch checkpoint file corruption on OS crash

2024-05-01 Thread Jun Rao (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-16541?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=17842783#comment-17842783 ] Jun Rao commented on KAFKA-16541: - [~ocadaruma] : Will you be able to work on this soon? The 3.8.0 code

Re: [PR] KAFKA-16452: Bound high-watermark offset to range between LLSO and LEO [kafka]

2024-05-01 Thread via GitHub
junrao commented on code in PR #15634: URL: https://github.com/apache/kafka/pull/15634#discussion_r1586840549 ## core/src/main/scala/kafka/log/UnifiedLog.scala: ## @@ -282,15 +282,15 @@ class UnifiedLog(@volatile var logStartOffset: Long, /** * Update high watermark

Re: [PR] KAFKA-16452: Don't throw OOORE when converting the offset to metadata [kafka]

2024-05-01 Thread via GitHub
junrao commented on code in PR #15825: URL: https://github.com/apache/kafka/pull/15825#discussion_r1586835182 ## storage/src/main/java/org/apache/kafka/storage/internals/log/LogOffsetMetadata.java: ## @@ -28,6 +28,7 @@ public final class LogOffsetMetadata { //TODO

Re: [PR] KAFKA-13328, KAFKA-13329 (2): Add custom preflight validation support for connector header, key, and value converters [kafka]

2024-05-01 Thread via GitHub
C0urante commented on code in PR #14309: URL: https://github.com/apache/kafka/pull/14309#discussion_r1586591019 ## connect/runtime/src/main/java/org/apache/kafka/connect/runtime/AbstractHerder.java: ## @@ -655,27 +811,38 @@ private static ConfigInfos

Re: [PR] KAFKA-16207; KRaft's internal log listener to update voter set [kafka]

2024-05-01 Thread via GitHub
jsancio commented on code in PR #15671: URL: https://github.com/apache/kafka/pull/15671#discussion_r1586828276 ## raft/src/main/java/org/apache/kafka/raft/QuorumConfig.java: ## @@ -206,8 +151,16 @@ private static Integer parseVoterId(String idString) { } } -

Re: [PR] KAFKA-16207; KRaft's internal log listener to update voter set [kafka]

2024-05-01 Thread via GitHub
jsancio commented on code in PR #15671: URL: https://github.com/apache/kafka/pull/15671#discussion_r1586825107 ## core/src/main/scala/kafka/raft/RaftManager.scala: ## @@ -181,20 +181,12 @@ class KafkaRaftManager[T]( private val clientDriver = new

Re: [PR] KAFKA-16207; KRaft's internal log listener to update voter set [kafka]

2024-05-01 Thread via GitHub
jsancio commented on code in PR #15671: URL: https://github.com/apache/kafka/pull/15671#discussion_r1586802072 ## raft/src/main/java/org/apache/kafka/raft/internals/History.java: ## @@ -0,0 +1,119 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one or more + *

Re: [PR] KAFKA-15974: Enforce that event processing respects user-provided timeout [kafka]

2024-05-01 Thread via GitHub
kirktrue commented on PR #15640: URL: https://github.com/apache/kafka/pull/15640#issuecomment-2089079055 > > > Here I have a comment, I could not put at the right location in the code: > > > On line 1362, in commitSync() the consumer waits on the commitFuture with a timer. I think, it

Re: [PR] KAFKA-16207; KRaft's internal log listener to update voter set [kafka]

2024-05-01 Thread via GitHub
jsancio commented on code in PR #15671: URL: https://github.com/apache/kafka/pull/15671#discussion_r1586782977 ## raft/src/test/java/org/apache/kafka/raft/internals/VoterSetHistoryTest.java: ## @@ -0,0 +1,167 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one

Re: [PR] KAFKA-16207; KRaft's internal log listener to update voter set [kafka]

2024-05-01 Thread via GitHub
jsancio commented on code in PR #15671: URL: https://github.com/apache/kafka/pull/15671#discussion_r1586765460 ## raft/src/test/java/org/apache/kafka/raft/internals/KRaftControlRecordStateMachineTest.java: ## @@ -0,0 +1,375 @@ +/* + * Licensed to the Apache Software Foundation

Re: [PR] KAFKA-16207; KRaft's internal log listener to update voter set [kafka]

2024-05-01 Thread via GitHub
jsancio commented on code in PR #15671: URL: https://github.com/apache/kafka/pull/15671#discussion_r1586760874 ## raft/src/main/java/org/apache/kafka/snapshot/RecordsSnapshotWriter.java: ## @@ -216,4 +140,132 @@ private void appendBatches(List> batches) {

[jira] [Commented] (KAFKA-16514) Kafka Streams: stream.close(CloseOptions) does not respect options.leaveGroup flag.

2024-05-01 Thread Lianet Magrans (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-16514?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=17842765#comment-17842765 ] Lianet Magrans commented on KAFKA-16514: Just to answer the question above: {quote}what is the

Re: [PR] KAFKA-16207; KRaft's internal log listener to update voter set [kafka]

2024-05-01 Thread via GitHub
jsancio commented on code in PR #15671: URL: https://github.com/apache/kafka/pull/15671#discussion_r1586731329 ## raft/src/main/java/org/apache/kafka/raft/internals/VoterSet.java: ## @@ -0,0 +1,327 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one or more +

Re: [PR] KAFKA-15974: Enforce that event processing respects user-provided timeout [kafka]

2024-05-01 Thread via GitHub
kirktrue commented on code in PR #15640: URL: https://github.com/apache/kafka/pull/15640#discussion_r1586632167 ## clients/src/main/java/org/apache/kafka/clients/consumer/internals/AsyncKafkaConsumer.java: ## @@ -1169,8 +1129,7 @@ private Map beginningOrEndOffset(Collection

Re: [PR] KAFKA-15974: Enforce that event processing respects user-provided timeout [kafka]

2024-05-01 Thread via GitHub
kirktrue commented on code in PR #15640: URL: https://github.com/apache/kafka/pull/15640#discussion_r1586626516 ## clients/src/test/java/org/apache/kafka/clients/consumer/KafkaConsumerTest.java: ## @@ -987,6 +987,7 @@ public void testResetUsingAutoResetPolicy(GroupProtocol

Re: [PR] KAFKA-15974: Enforce that event processing respects user-provided timeout [kafka]

2024-05-01 Thread via GitHub
kirktrue commented on code in PR #15640: URL: https://github.com/apache/kafka/pull/15640#discussion_r1586622390 ## clients/src/main/java/org/apache/kafka/clients/consumer/internals/AsyncKafkaConsumer.java: ## @@ -1273,6 +1228,22 @@ private void close(Duration timeout, boolean

Re: [PR] KAFKA-15974: Enforce that event processing respects user-provided timeout [kafka]

2024-05-01 Thread via GitHub
kirktrue commented on code in PR #15640: URL: https://github.com/apache/kafka/pull/15640#discussion_r1586620323 ## clients/src/main/java/org/apache/kafka/clients/consumer/internals/events/CompletableEvent.java: ## @@ -16,9 +16,118 @@ */ package

Re: [PR] KAFKA-15974: Enforce that event processing respects user-provided timeout [kafka]

2024-05-01 Thread via GitHub
kirktrue commented on code in PR #15640: URL: https://github.com/apache/kafka/pull/15640#discussion_r1586620323 ## clients/src/main/java/org/apache/kafka/clients/consumer/internals/events/CompletableEvent.java: ## @@ -16,9 +16,118 @@ */ package

Re: [PR] KAFKA-15974: Enforce that event processing respects user-provided timeout [kafka]

2024-05-01 Thread via GitHub
kirktrue commented on code in PR #15640: URL: https://github.com/apache/kafka/pull/15640#discussion_r1586607598 ## clients/src/main/java/org/apache/kafka/clients/consumer/internals/events/CommitEvent.java: ## @@ -30,12 +29,7 @@ public abstract class CommitEvent extends

Re: [PR] KAFKA-15974: Enforce that event processing respects user-provided timeout [kafka]

2024-05-01 Thread via GitHub
kirktrue commented on code in PR #15640: URL: https://github.com/apache/kafka/pull/15640#discussion_r1586603709 ## clients/src/main/java/org/apache/kafka/clients/consumer/internals/ConsumerNetworkThread.java: ## @@ -273,9 +310,18 @@ void cleanup() {

Re: [PR] KAFKA-13328, KAFKA-13329 (2): Add custom preflight validation support for connector header, key, and value converters [kafka]

2024-05-01 Thread via GitHub
C0urante commented on code in PR #14309: URL: https://github.com/apache/kafka/pull/14309#discussion_r1586591574 ## connect/runtime/src/main/java/org/apache/kafka/connect/runtime/AbstractHerder.java: ## @@ -655,27 +811,38 @@ private static ConfigInfos

Re: [PR] KAFKA-13328, KAFKA-13329 (2): Add custom preflight validation support for connector header, key, and value converters [kafka]

2024-05-01 Thread via GitHub
C0urante commented on code in PR #14309: URL: https://github.com/apache/kafka/pull/14309#discussion_r1586591136 ## connect/runtime/src/main/java/org/apache/kafka/connect/runtime/AbstractHerder.java: ## @@ -392,6 +399,146 @@ protected Map

Re: [PR] KAFKA-13328, KAFKA-13329 (2): Add custom preflight validation support for connector header, key, and value converters [kafka]

2024-05-01 Thread via GitHub
C0urante commented on code in PR #14309: URL: https://github.com/apache/kafka/pull/14309#discussion_r1586590857 ## connect/runtime/src/main/java/org/apache/kafka/connect/runtime/AbstractHerder.java: ## @@ -562,8 +709,13 @@ ConfigInfos validateConnectorConfig(

[jira] [Commented] (KAFKA-16022) AsyncKafkaConsumer sometimes complains “No current assignment for partition {}”

2024-05-01 Thread Philip Nee (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-16022?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=17842730#comment-17842730 ] Philip Nee commented on KAFKA-16022: hi [~phuctran] - I believe this came up during integration

Re: [PR] KAFKA-15974: Enforce that event processing respects user-provided timeout [kafka]

2024-05-01 Thread via GitHub
kirktrue commented on code in PR #15640: URL: https://github.com/apache/kafka/pull/15640#discussion_r1586559943 ## clients/src/main/java/org/apache/kafka/clients/consumer/internals/ConsumerNetworkThread.java: ## @@ -273,9 +310,18 @@ void cleanup() {

Re: [PR] KAFKA-15974: Enforce that event processing respects user-provided timeout [kafka]

2024-05-01 Thread via GitHub
kirktrue commented on code in PR #15640: URL: https://github.com/apache/kafka/pull/15640#discussion_r1586557142 ## clients/src/main/java/org/apache/kafka/clients/consumer/internals/AsyncKafkaConsumer.java: ## @@ -1892,13 +1897,13 @@ private void subscribeInternal(Collection

Re: [PR] KAFKA-15974: Enforce that event processing respects user-provided timeout [kafka]

2024-05-01 Thread via GitHub
kirktrue commented on code in PR #15640: URL: https://github.com/apache/kafka/pull/15640#discussion_r1586557142 ## clients/src/main/java/org/apache/kafka/clients/consumer/internals/AsyncKafkaConsumer.java: ## @@ -1892,13 +1897,13 @@ private void subscribeInternal(Collection

Re: [PR] KAFKA-16027: MINOR Refactor MetadataTest#testUpdatePartitionLeadership [kafka]

2024-05-01 Thread via GitHub
chia7712 commented on PR #15055: URL: https://github.com/apache/kafka/pull/15055#issuecomment-2088823785 > I have the correct file still saved on my new forced update but I can't link it to this issue. I don't want to revert because I accidently pushed all the commits from the past few

[jira] [Commented] (KAFKA-16027) Refactor MetadataTest#testUpdatePartitionLeadership

2024-05-01 Thread Alexander Aghili (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-16027?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=17842727#comment-17842727 ] Alexander Aghili commented on KAFKA-16027: -- Hi [~johnnyhsu], I have added it the feature but

Re: [PR] KAFKA-16207; KRaft's internal log listener to update voter set [kafka]

2024-05-01 Thread via GitHub
jsancio commented on code in PR #15671: URL: https://github.com/apache/kafka/pull/15671#discussion_r1586553978 ## raft/src/main/java/org/apache/kafka/raft/internals/VoterSet.java: ## @@ -0,0 +1,327 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one or more +

Re: [PR] KAFKA-16027: MINOR Refactor MetadataTest#testUpdatePartitionLeadership [kafka]

2024-05-01 Thread via GitHub
Alexander-Aghili commented on PR #15055: URL: https://github.com/apache/kafka/pull/15055#issuecomment-2088813846 I have the correct file still saved on my new forced update but I can't link it to this issue. I don't want to revert because I accidently pushed all the commits from the past

Re: [PR] KAFKA-15974: Enforce that event processing respects user-provided timeout [kafka]

2024-05-01 Thread via GitHub
kirktrue commented on code in PR #15640: URL: https://github.com/apache/kafka/pull/15640#discussion_r1586542049 ## clients/src/main/java/org/apache/kafka/clients/consumer/internals/AsyncKafkaConsumer.java: ## @@ -1853,6 +1824,40 @@ private void subscribeInternal(Collection

Re: [PR] KAFKA-15974: Enforce that event processing respects user-provided timeout [kafka]

2024-05-01 Thread via GitHub
kirktrue commented on code in PR #15640: URL: https://github.com/apache/kafka/pull/15640#discussion_r1586539952 ## clients/src/main/java/org/apache/kafka/clients/consumer/internals/events/CompletableEventReaper.java: ## @@ -0,0 +1,150 @@ +/* + * Licensed to the Apache Software

Re: [PR] KAFKA-15974: Enforce that event processing respects user-provided timeout [kafka]

2024-05-01 Thread via GitHub
kirktrue commented on code in PR #15640: URL: https://github.com/apache/kafka/pull/15640#discussion_r1586537358 ## clients/src/main/java/org/apache/kafka/clients/consumer/internals/events/CompletableEventReaper.java: ## @@ -0,0 +1,150 @@ +/* + * Licensed to the Apache Software

[jira] [Commented] (KAFKA-16223) Replace EasyMock and PowerMock with Mockito for KafkaConfigBackingStoreTest

2024-05-01 Thread Chia-Ping Tsai (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-16223?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=17842724#comment-17842724 ] Chia-Ping Tsai commented on KAFKA-16223: {quote} I have another set of tests already migrated

Re: [PR] KAFKA-16207; KRaft's internal log listener to update voter set [kafka]

2024-05-01 Thread via GitHub
jsancio commented on code in PR #15671: URL: https://github.com/apache/kafka/pull/15671#discussion_r1586526016 ## raft/src/main/java/org/apache/kafka/raft/ReplicatedLog.java: ## @@ -261,7 +260,7 @@ default long truncateToEndOffset(OffsetAndEpoch endOffset) { * @param

Re: [PR] KAFKA-15974: Enforce that event processing respects user-provided timeout [kafka]

2024-05-01 Thread via GitHub
kirktrue commented on code in PR #15640: URL: https://github.com/apache/kafka/pull/15640#discussion_r1586524076 ## clients/src/main/java/org/apache/kafka/clients/consumer/internals/events/CompletableEventReaper.java: ## @@ -0,0 +1,150 @@ +/* + * Licensed to the Apache Software

[PR] [WIP] KAFKA-16027: refactor MetadataTest [kafka]

2024-05-01 Thread via GitHub
johnnychhsu opened a new pull request, #15842: URL: https://github.com/apache/kafka/pull/15842 Jira: https://issues.apache.org/jira/browse/KAFKA-16027 ### Committer Checklist (excluded from commit message) - [ ] Verify design and implementation - [ ] Verify test coverage and CI

[PR] KAFKA-16223 Replace EasyMock/PowerMock with Mockito for KafkaConfigBackingStoreTest (2/3) [kafka]

2024-05-01 Thread via GitHub
hgeraldino opened a new pull request, #15841: URL: https://github.com/apache/kafka/pull/15841 This is the last remaining Kafka Connect test that needs migration from PowerMock/EasyMock to Mockito. Previous PR: https://github.com/apache/kafka/pull/15520 As usual, I Iook forward

Re: [PR] KAFKA-15309: Add custom error handler to Producer [kafka]

2024-05-01 Thread via GitHub
aliehsaeedii closed pull request #15731: KAFKA-15309: Add custom error handler to Producer URL: https://github.com/apache/kafka/pull/15731 -- This is an automated message from the Apache Git Service. To respond to the message, please log on to GitHub and use the URL above to go to the

Re: [PR] KAFKA-15974: Enforce that event processing respects user-provided timeout [kafka]

2024-05-01 Thread via GitHub
kirktrue commented on code in PR #15640: URL: https://github.com/apache/kafka/pull/15640#discussion_r1586499310 ## clients/src/main/java/org/apache/kafka/clients/consumer/internals/events/CompletableEventReaper.java: ## @@ -0,0 +1,150 @@ +/* + * Licensed to the Apache Software

[jira] [Commented] (KAFKA-16223) Replace EasyMock and PowerMock with Mockito for KafkaConfigBackingStoreTest

2024-05-01 Thread Hector Geraldino (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-16223?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=17842717#comment-17842717 ] Hector Geraldino commented on KAFKA-16223: -- Thanks [~chia7712] [~cmukka20] for following up on

[jira] [Assigned] (KAFKA-16652) add unit test for ClusterTemplate offering zero ClusterConfig

2024-05-01 Thread TaiJuWu (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-16652?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] TaiJuWu reassigned KAFKA-16652: --- Assignee: TaiJuWu (was: Chia-Ping Tsai) > add unit test for ClusterTemplate offering zero

Re: [PR] KAFKA-16207; KRaft's internal log listener to update voter set [kafka]

2024-05-01 Thread via GitHub
jsancio commented on code in PR #15671: URL: https://github.com/apache/kafka/pull/15671#discussion_r1586489784 ## clients/src/main/java/org/apache/kafka/common/record/MemoryRecords.java: ## @@ -807,4 +809,62 @@ private static void writeSnapshotFooterRecord(

Re: [PR] KAFKA-16614:Disallow @ClusterTemplate("") [kafka]

2024-05-01 Thread via GitHub
TaiJuWu commented on code in PR #15800: URL: https://github.com/apache/kafka/pull/15800#discussion_r1586484285 ## core/src/test/java/kafka/test/junit/ClusterTestExtensionsUnitTest.java: ## @@ -0,0 +1,45 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one or

[jira] [Updated] (KAFKA-16272) Update connect_distributed_test.py to support KIP-848’s group protocol config

2024-05-01 Thread Kirk True (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-16272?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Kirk True updated KAFKA-16272: -- Fix Version/s: 3.8.0 (was: 4.0.0) > Update connect_distributed_test.py to

[jira] [Updated] (KAFKA-16272) Update connect_distributed_test.py to support KIP-848’s group protocol config

2024-05-01 Thread Kirk True (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-16272?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Kirk True updated KAFKA-16272: -- Priority: Major (was: Blocker) > Update connect_distributed_test.py to support KIP-848’s group

[jira] [Updated] (KAFKA-15561) Client support for new SubscriptionPattern based subscription

2024-05-01 Thread Kirk True (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-15561?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Kirk True updated KAFKA-15561: -- Fix Version/s: 3.8.0 (was: 4.0.0) > Client support for new SubscriptionPattern

  1   2   >