[jira] [Updated] (KAFKA-14842) MirrorCheckpointTask can reduce the rpc calls of "listConsumerGroupOffsets(group)" of irrelevant groups at each poll

2023-03-23 Thread hudeqi (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-14842?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] hudeqi updated KAFKA-14842: --- Parent: (was: KAFKA-14824) Issue Type: Improvement (was: Sub-task) > MirrorCheckpointTask can

[jira] [Updated] (KAFKA-14842) MirrorCheckpointTask can reduce the rpc calls of "listConsumerGroupOffsets(group)" of irrelevant groups at each poll

2023-03-23 Thread hudeqi (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-14842?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] hudeqi updated KAFKA-14842: --- Description: sorry, wrong related. > MirrorCheckpointTask can reduce the rpc calls of >

[jira] [Created] (KAFKA-14842) MirrorCheckpointTask can reduce the rpc calls of "listConsumerGroupOffsets(group)" of irrelevant groups at each poll

2023-03-23 Thread hudeqi (Jira)
hudeqi created KAFKA-14842: -- Summary: MirrorCheckpointTask can reduce the rpc calls of "listConsumerGroupOffsets(group)" of irrelevant groups at each poll Key: KAFKA-14842 URL:

[jira] [Commented] (KAFKA-14837) The MirrorCheckPointConnector of MM2 will rebalance frequently, when the source cluster group is many more and changes frequently (but the list of configured synchrono

2023-03-23 Thread hudeqi (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-14837?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=17704420#comment-17704420 ] hudeqi commented on KAFKA-14837: Thanks for the reply! Increasing the parameter

[GitHub] [kafka] guozhangwang commented on a diff in pull request #11433: KAFKA-13295: Avoiding Transation timeouts arising due to long restora…

2023-03-23 Thread via GitHub
guozhangwang commented on code in PR #11433: URL: https://github.com/apache/kafka/pull/11433#discussion_r1147064683 ## streams/src/main/java/org/apache/kafka/streams/processor/internals/TaskManager.java: ## @@ -342,7 +379,18 @@ public void handleAssignment(final Map>

[GitHub] [kafka] vcrfxia commented on a diff in pull request #13444: KAFKA-14491: [21/N] Docs updates for versioned state stores

2023-03-23 Thread via GitHub
vcrfxia commented on code in PR #13444: URL: https://github.com/apache/kafka/pull/13444#discussion_r1147020240 ## streams/src/main/java/org/apache/kafka/streams/state/Stores.java: ## @@ -166,6 +248,11 @@ public String metricsScope() { * This store supplier can be passed

[GitHub] [kafka] vcrfxia opened a new pull request, #13444: KAFKA-14491: [21/N] Docs updates for versioned state stores

2023-03-23 Thread via GitHub
vcrfxia opened a new pull request, #13444: URL: https://github.com/apache/kafka/pull/13444 (This PR is stacked on https://github.com/apache/kafka/pull/13442. The first commit does not need to be reviewed separately.) This PR adds docs for the versioned stores introduced in

[jira] [Updated] (KAFKA-14841) Call ConsumerRebalanceListener when MockConsumer rebalanced

2023-03-23 Thread Daniel Scanteianu (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-14841?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Daniel Scanteianu updated KAFKA-14841: -- Description: Currently, passing a custom ConsumerRebalanceListener to mock client is

[jira] [Created] (KAFKA-14841) Call ConsumerRebalanceListener when MockConsumer rebalanced

2023-03-23 Thread Daniel Scanteianu (Jira)
Daniel Scanteianu created KAFKA-14841: - Summary: Call ConsumerRebalanceListener when MockConsumer rebalanced Key: KAFKA-14841 URL: https://issues.apache.org/jira/browse/KAFKA-14841 Project: Kafka

[GitHub] [kafka] philipnee commented on a diff in pull request #13380: KAFKA-14468: Committed API

2023-03-23 Thread via GitHub
philipnee commented on code in PR #13380: URL: https://github.com/apache/kafka/pull/13380#discussion_r1146979021 ## clients/src/main/java/org/apache/kafka/clients/consumer/internals/CommitRequestManager.java: ## @@ -137,31 +166,21 @@ CompletableFuture sendAutoCommit(final Map

[GitHub] [kafka] kirktrue commented on a diff in pull request #13425: KAFKA-14365: Extract common logic from Fetcher

2023-03-23 Thread via GitHub
kirktrue commented on code in PR #13425: URL: https://github.com/apache/kafka/pull/13425#discussion_r1146976691 ## clients/src/main/java/org/apache/kafka/clients/consumer/internals/AbstractFetch.java: ## @@ -0,0 +1,791 @@ +/* + * Licensed to the Apache Software Foundation (ASF)

[GitHub] [kafka] kirktrue commented on a diff in pull request #13425: KAFKA-14365: Extract common logic from Fetcher

2023-03-23 Thread via GitHub
kirktrue commented on code in PR #13425: URL: https://github.com/apache/kafka/pull/13425#discussion_r1146975671 ## clients/src/main/java/org/apache/kafka/clients/consumer/internals/AbstractFetch.java: ## @@ -0,0 +1,791 @@ +/* + * Licensed to the Apache Software Foundation (ASF)

[GitHub] [kafka] kirktrue commented on a diff in pull request #13425: KAFKA-14365: Extract common logic from Fetcher

2023-03-23 Thread via GitHub
kirktrue commented on code in PR #13425: URL: https://github.com/apache/kafka/pull/13425#discussion_r1146975156 ## clients/src/main/java/org/apache/kafka/clients/consumer/internals/AbstractFetch.java: ## @@ -0,0 +1,791 @@ +/* + * Licensed to the Apache Software Foundation (ASF)

[GitHub] [kafka] kirktrue commented on a diff in pull request #13425: KAFKA-14365: Extract common logic from Fetcher

2023-03-23 Thread via GitHub
kirktrue commented on code in PR #13425: URL: https://github.com/apache/kafka/pull/13425#discussion_r1146975156 ## clients/src/main/java/org/apache/kafka/clients/consumer/internals/AbstractFetch.java: ## @@ -0,0 +1,791 @@ +/* + * Licensed to the Apache Software Foundation (ASF)

[GitHub] [kafka] philipnee commented on pull request #13380: KAFKA-14468: Committed API

2023-03-23 Thread via GitHub
philipnee commented on PR #13380: URL: https://github.com/apache/kafka/pull/13380#issuecomment-1482045502 Hey @guozhangwang sorry for going back and forth on this, I'll add some test cases to cover your comments. -- This is an automated message from the Apache Git Service. To respond to

[GitHub] [kafka] philipnee commented on a diff in pull request #13380: KAFKA-14468: Committed API

2023-03-23 Thread via GitHub
philipnee commented on code in PR #13380: URL: https://github.com/apache/kafka/pull/13380#discussion_r1146971049 ## clients/src/main/java/org/apache/kafka/clients/consumer/internals/CommitRequestManager.java: ## @@ -137,31 +166,21 @@ CompletableFuture sendAutoCommit(final Map

[GitHub] [kafka] philipnee commented on a diff in pull request #13380: KAFKA-14468: Committed API

2023-03-23 Thread via GitHub
philipnee commented on code in PR #13380: URL: https://github.com/apache/kafka/pull/13380#discussion_r1146969176 ## clients/src/main/java/org/apache/kafka/clients/consumer/internals/CommitRequestManager.java: ## @@ -207,6 +226,232 @@ public NetworkClientDelegate.UnsentRequest

[GitHub] [kafka] philipnee commented on a diff in pull request #13380: KAFKA-14468: Committed API

2023-03-23 Thread via GitHub
philipnee commented on code in PR #13380: URL: https://github.com/apache/kafka/pull/13380#discussion_r1146967451 ## clients/src/main/java/org/apache/kafka/clients/consumer/internals/CommitRequestManager.java: ## @@ -207,6 +226,232 @@ public NetworkClientDelegate.UnsentRequest

[GitHub] [kafka] kirktrue commented on a diff in pull request #13425: KAFKA-14365: Extract common logic from Fetcher

2023-03-23 Thread via GitHub
kirktrue commented on code in PR #13425: URL: https://github.com/apache/kafka/pull/13425#discussion_r1146964967 ## clients/src/main/java/org/apache/kafka/clients/consumer/internals/Fetcher.java: ## @@ -172,606 +69,41 @@ public boolean hasAvailableFetches() { * @return

[GitHub] [kafka] guozhangwang commented on a diff in pull request #13425: KAFKA-14365: Extract common logic from Fetcher

2023-03-23 Thread via GitHub
guozhangwang commented on code in PR #13425: URL: https://github.com/apache/kafka/pull/13425#discussion_r1146964838 ## clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerConfig.java: ## @@ -154,7 +155,8 @@ public class ConsumerConfig extends AbstractConfig {

[GitHub] [kafka] guozhangwang commented on pull request #13380: KAFKA-14468: Committed API

2023-03-23 Thread via GitHub
guozhangwang commented on PR #13380: URL: https://github.com/apache/kafka/pull/13380#issuecomment-1482028180 @philipnee I made a third pass on the PR, but it seems we still have a few issues not fixed: 1) https://github.com/apache/kafka/pull/13380#discussion_r1146961554, 2)

[GitHub] [kafka] guozhangwang commented on a diff in pull request #13380: KAFKA-14468: Committed API

2023-03-23 Thread via GitHub
guozhangwang commented on code in PR #13380: URL: https://github.com/apache/kafka/pull/13380#discussion_r1146963166 ## clients/src/main/java/org/apache/kafka/clients/consumer/internals/CommitRequestManager.java: ## @@ -137,31 +166,21 @@ CompletableFuture sendAutoCommit(final

[GitHub] [kafka] guozhangwang commented on a diff in pull request #13380: KAFKA-14468: Committed API

2023-03-23 Thread via GitHub
guozhangwang commented on code in PR #13380: URL: https://github.com/apache/kafka/pull/13380#discussion_r1146961554 ## clients/src/main/java/org/apache/kafka/clients/consumer/internals/CommitRequestManager.java: ## @@ -207,6 +226,232 @@ public

[GitHub] [kafka] kirktrue commented on a diff in pull request #13425: KAFKA-14365: Extract common logic from Fetcher

2023-03-23 Thread via GitHub
kirktrue commented on code in PR #13425: URL: https://github.com/apache/kafka/pull/13425#discussion_r1146961449 ## clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerConfig.java: ## @@ -154,7 +155,8 @@ public class ConsumerConfig extends AbstractConfig { *

[GitHub] [kafka] guozhangwang commented on a diff in pull request #13380: KAFKA-14468: Committed API

2023-03-23 Thread via GitHub
guozhangwang commented on code in PR #13380: URL: https://github.com/apache/kafka/pull/13380#discussion_r1146959499 ## clients/src/main/java/org/apache/kafka/clients/consumer/internals/CommitRequestManager.java: ## @@ -207,6 +226,232 @@ public

[GitHub] [kafka] guozhangwang commented on a diff in pull request #13425: KAFKA-14365: Extract common logic from Fetcher

2023-03-23 Thread via GitHub
guozhangwang commented on code in PR #13425: URL: https://github.com/apache/kafka/pull/13425#discussion_r1146949108 ## clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerConfig.java: ## @@ -154,7 +155,8 @@ public class ConsumerConfig extends AbstractConfig {

[GitHub] [kafka] jolshan commented on a diff in pull request #13391: KAFKA-14561: Improve transactions experience for older clients by ensuring ongoing transaction

2023-03-23 Thread via GitHub
jolshan commented on code in PR #13391: URL: https://github.com/apache/kafka/pull/13391#discussion_r1146951880 ## core/src/main/scala/kafka/server/AddPartitionsToTxnManager.scala: ## @@ -0,0 +1,173 @@ +/** + * Licensed to the Apache Software Foundation (ASF) under one or more +

[GitHub] [kafka] junrao commented on a diff in pull request #13391: KAFKA-14561: Improve transactions experience for older clients by ensuring ongoing transaction

2023-03-23 Thread via GitHub
junrao commented on code in PR #13391: URL: https://github.com/apache/kafka/pull/13391#discussion_r1146944915 ## core/src/main/scala/kafka/server/KafkaRequestHandler.scala: ## @@ -35,6 +36,40 @@ trait ApiRequestHandler { def handle(request: RequestChannel.Request,

[GitHub] [kafka] SpacRocket commented on pull request #13382: KAFKA-14722: Make BooleanSerde public

2023-03-23 Thread via GitHub
SpacRocket commented on PR #13382: URL: https://github.com/apache/kafka/pull/13382#issuecomment-1481980186 Apple M1: JDK 17, Scala 2.13 testDeleteInternalTopic [1st test] **(2/2 tests passed)** testDescribeQuorumStatusSuccesful [2nd test] **(6/6 tests passed)**

[GitHub] [kafka] cmccabe commented on a diff in pull request #13438: KAFKA-14835: Create ControllerServerMetricsPublisher

2023-03-23 Thread via GitHub
cmccabe commented on code in PR #13438: URL: https://github.com/apache/kafka/pull/13438#discussion_r1146927689 ## core/src/main/scala/kafka/server/ControllerServer.scala: ## @@ -104,13 +106,15 @@ class ControllerServer( val socketServerFirstBoundPortFuture = new

[GitHub] [kafka] cmccabe commented on a diff in pull request #13438: KAFKA-14835: Create ControllerServerMetricsPublisher

2023-03-23 Thread via GitHub
cmccabe commented on code in PR #13438: URL: https://github.com/apache/kafka/pull/13438#discussion_r1146914196 ## metadata/src/main/java/org/apache/kafka/controller/metrics/ControllerServerMetrics.java: ## @@ -0,0 +1,211 @@ +/* + * Licensed to the Apache Software Foundation

[GitHub] [kafka] cmccabe commented on a diff in pull request #13438: KAFKA-14835: Create ControllerServerMetricsPublisher

2023-03-23 Thread via GitHub
cmccabe commented on code in PR #13438: URL: https://github.com/apache/kafka/pull/13438#discussion_r1146909450 ## metadata/src/main/java/org/apache/kafka/controller/metrics/ControllerServerMetrics.java: ## @@ -0,0 +1,211 @@ +/* + * Licensed to the Apache Software Foundation

[GitHub] [kafka] cmccabe commented on a diff in pull request #13438: KAFKA-14835: Create ControllerServerMetricsPublisher

2023-03-23 Thread via GitHub
cmccabe commented on code in PR #13438: URL: https://github.com/apache/kafka/pull/13438#discussion_r1146908247 ## metadata/src/main/java/org/apache/kafka/controller/metrics/ControllerServerMetricsPublisher.java: ## @@ -0,0 +1,156 @@ +/* + * Licensed to the Apache Software

[GitHub] [kafka] cmccabe commented on a diff in pull request #13438: KAFKA-14835: Create ControllerServerMetricsPublisher

2023-03-23 Thread via GitHub
cmccabe commented on code in PR #13438: URL: https://github.com/apache/kafka/pull/13438#discussion_r1146902273 ## metadata/src/main/java/org/apache/kafka/controller/metrics/ControllerMetricsChanges.java: ## @@ -0,0 +1,147 @@ +/* + * Licensed to the Apache Software Foundation

[jira] [Updated] (KAFKA-14776) Update SCRAM integration tests to run with KRaft

2023-03-23 Thread Proven Provenzano (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-14776?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Proven Provenzano updated KAFKA-14776: -- Fix Version/s: 3.5.0 > Update SCRAM integration tests to run with KRaft >

[jira] [Updated] (KAFKA-14765) Support SCRAM for brokers at bootstrap

2023-03-23 Thread Proven Provenzano (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-14765?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Proven Provenzano updated KAFKA-14765: -- Fix Version/s: 3.5.0 > Support SCRAM for brokers at bootstrap >

[jira] [Updated] (KAFKA-14084) Support SCRAM when using KRaft mode

2023-03-23 Thread Proven Provenzano (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-14084?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Proven Provenzano updated KAFKA-14084: -- Fix Version/s: 3.5.0 > Support SCRAM when using KRaft mode >

[GitHub] [kafka] cmccabe commented on a diff in pull request #13438: KAFKA-14835: Create ControllerServerMetricsPublisher

2023-03-23 Thread via GitHub
cmccabe commented on code in PR #13438: URL: https://github.com/apache/kafka/pull/13438#discussion_r1146899679 ## core/src/main/scala/kafka/server/ControllerServer.scala: ## @@ -366,6 +376,8 @@ class ControllerServer( // Ensure that we're not the Raft leader prior to

[GitHub] [kafka] pprovenzano commented on a diff in pull request #13374: KAFKA-14765 and KAFKA-14776: Support for SCRAM at bootstrap with integration tests

2023-03-23 Thread via GitHub
pprovenzano commented on code in PR #13374: URL: https://github.com/apache/kafka/pull/13374#discussion_r1146871469 ## core/src/main/scala/kafka/tools/StorageTool.scala: ## @@ -128,6 +152,108 @@ object StorageTool extends Logging { .getOrElse(defaultValue) } + def

[GitHub] [kafka] jolshan commented on a diff in pull request #13391: KAFKA-14561: Improve transactions experience for older clients by ensuring ongoing transaction

2023-03-23 Thread via GitHub
jolshan commented on code in PR #13391: URL: https://github.com/apache/kafka/pull/13391#discussion_r1146821982 ## core/src/main/scala/kafka/server/AddPartitionsToTxnManager.scala: ## @@ -0,0 +1,173 @@ +/** + * Licensed to the Apache Software Foundation (ASF) under one or more +

[GitHub] [kafka] YiDing-Duke commented on a diff in pull request #13391: KAFKA-14561: Improve transactions experience for older clients by ensuring ongoing transaction

2023-03-23 Thread via GitHub
YiDing-Duke commented on code in PR #13391: URL: https://github.com/apache/kafka/pull/13391#discussion_r1146789680 ## core/src/main/scala/kafka/server/AddPartitionsToTxnManager.scala: ## @@ -0,0 +1,173 @@ +/** + * Licensed to the Apache Software Foundation (ASF) under one or

[GitHub] [kafka] chia7712 commented on pull request #13407: KAFKA-14805 KRaft controller supports pre-migration mode

2023-03-23 Thread via GitHub
chia7712 commented on PR #13407: URL: https://github.com/apache/kafka/pull/13407#issuecomment-1481835395 @mumrah thanks for the response. I raised the question because 3.5.0 will get released on 4/26. The bugs happen when testing Zk migration so it would be nice to get fix into 3.5.0

[GitHub] [kafka] rreddy-22 opened a new pull request, #13443: KAFKA-14514: Add Server Side Sticky Range Assignor (KIP-848)

2023-03-23 Thread via GitHub
rreddy-22 opened a new pull request, #13443: URL: https://github.com/apache/kafka/pull/13443 **Part of KIP-848** As a part of the new protocol, there will be assignors on the server side as well to take some load off the client. The group coordinator either directly computes the new

[GitHub] [kafka] mumrah commented on pull request #13407: KAFKA-14805 KRaft controller supports pre-migration mode

2023-03-23 Thread via GitHub
mumrah commented on PR #13407: URL: https://github.com/apache/kafka/pull/13407#issuecomment-1481804340 @chia7712 it's quite possible that #13405 is another source of failures for ZkMigrationIntegrationTest#testDualWrite. This patch will fix the main source of failures that I've seen which

[jira] [Created] (KAFKA-14840) Handle KRaft snapshots in dual-write mode

2023-03-23 Thread David Arthur (Jira)
David Arthur created KAFKA-14840: Summary: Handle KRaft snapshots in dual-write mode Key: KAFKA-14840 URL: https://issues.apache.org/jira/browse/KAFKA-14840 Project: Kafka Issue Type:

[jira] [Commented] (KAFKA-14016) Revoke more partitions than expected in Cooperative rebalance

2023-03-23 Thread tony mancill (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-14016?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=17704322#comment-17704322 ] tony mancill commented on KAFKA-14016: -- We are seeing this issue with 3.3.2.  Quoting from one of

[jira] [Commented] (KAFKA-13295) Long restoration times for new tasks can lead to transaction timeouts

2023-03-23 Thread Sagar Rao (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-13295?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=17704320#comment-17704320 ] Sagar Rao commented on KAFKA-13295: --- Thanks [~guozhang] , glad to know that! > Long restoration times

[jira] [Updated] (KAFKA-14016) Revoke more partitions than expected in Cooperative rebalance

2023-03-23 Thread tony mancill (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-14016?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] tony mancill updated KAFKA-14016: - Attachment: CooperativeStickyAssignorBugReproduction.java > Revoke more partitions than

[GitHub] [kafka] vcrfxia opened a new pull request, #13442: KAFKA-14491: [20/N] Add public-facing methods for versioned stores

2023-03-23 Thread via GitHub
vcrfxia opened a new pull request, #13442: URL: https://github.com/apache/kafka/pull/13442 (This PR should not be merged until after https://github.com/apache/kafka/pull/13364 and https://github.com/apache/kafka/pull/13431 are merged, so that the feature implementation may be fully

[GitHub] [kafka] wcarlson5 commented on pull request #13382: KAFKA-14722: Make BooleanSerde public

2023-03-23 Thread via GitHub
wcarlson5 commented on PR #13382: URL: https://github.com/apache/kafka/pull/13382#issuecomment-1481712837 ![image](https://user-images.githubusercontent.com/18128741/227315082-ba7ff83a-46b5-4117-bdd3-c93ebee19d32.png) @SpacRocket The tests don't look related. Can you run them locally?

[GitHub] [kafka] dimitarndimitrov commented on pull request #13432: KAFKA-14821 Implement the listOffsets API with AdminApiDriver

2023-03-23 Thread via GitHub
dimitarndimitrov commented on PR #13432: URL: https://github.com/apache/kafka/pull/13432#issuecomment-1481689669 After a rerun, the CI this time failed on `CoordinatorTest.testTaskRequestWithOldStartMsGetsUpdated` and

[GitHub] [kafka] chia7712 commented on pull request #13407: KAFKA-14805 KRaft controller supports pre-migration mode

2023-03-23 Thread via GitHub
chia7712 commented on PR #13407: URL: https://github.com/apache/kafka/pull/13407#issuecomment-1481642554 @mumrah thanks for this patch. pardon me that I don't dig in this patch before. Do you think the bug related to both #13405 and ZkMigrationIntegrationTest#testDualWrite is fixed by this

[GitHub] [kafka] cmccabe commented on pull request #13374: KAFKA-14765 and KAFKA-14776: Support for SCRAM at bootstrap with integration tests

2023-03-23 Thread via GitHub
cmccabe commented on PR #13374: URL: https://github.com/apache/kafka/pull/13374#issuecomment-1481634752 This seems to have broken `FeatureCommandUnitTest.testHandleDowngrade` , which is concerning since I don't think that tests SCRAM. -- This is an automated message from the Apache Git

[GitHub] [kafka] cmccabe commented on a diff in pull request #13374: KAFKA-14765 and KAFKA-14776: Support for SCRAM at bootstrap with integration tests

2023-03-23 Thread via GitHub
cmccabe commented on code in PR #13374: URL: https://github.com/apache/kafka/pull/13374#discussion_r1146574776 ## core/src/test/scala/unit/kafka/tools/StorageToolTest.scala: ## @@ -221,4 +224,78 @@ Found problem: assertThrows(classOf[IllegalArgumentException], () =>

[GitHub] [kafka] cmccabe commented on a diff in pull request #13374: KAFKA-14765 and KAFKA-14776: Support for SCRAM at bootstrap with integration tests

2023-03-23 Thread via GitHub
cmccabe commented on code in PR #13374: URL: https://github.com/apache/kafka/pull/13374#discussion_r1146573531 ## core/src/main/scala/kafka/tools/StorageTool.scala: ## @@ -128,6 +152,108 @@ object StorageTool extends Logging { .getOrElse(defaultValue) } + def

[GitHub] [kafka] chia7712 commented on pull request #13393: KAFKA-10244 An new java interface to replace 'kafka.common.MessageReader'

2023-03-23 Thread via GitHub
chia7712 commented on PR #13393: URL: https://github.com/apache/kafka/pull/13393#issuecomment-1481619154 @mimaison thanks for reviews. I have addressed all comments. please take a look, thanks! -- This is an automated message from the Apache Git Service. To respond to the message, please

[GitHub] [kafka] cmccabe commented on a diff in pull request #13345: KAFKA-13020; Implement reading Snapshot log append timestamp

2023-03-23 Thread via GitHub
cmccabe commented on code in PR #13345: URL: https://github.com/apache/kafka/pull/13345#discussion_r1146558110 ## raft/src/main/java/org/apache/kafka/raft/ControlRecord.java: ## @@ -0,0 +1,104 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one or more + *

[GitHub] [kafka] cmccabe commented on a diff in pull request #13345: KAFKA-13020; Implement reading Snapshot log append timestamp

2023-03-23 Thread via GitHub
cmccabe commented on code in PR #13345: URL: https://github.com/apache/kafka/pull/13345#discussion_r1146558110 ## raft/src/main/java/org/apache/kafka/raft/ControlRecord.java: ## @@ -0,0 +1,104 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one or more + *

[GitHub] [kafka] cmccabe commented on a diff in pull request #13345: KAFKA-13020; Implement reading Snapshot log append timestamp

2023-03-23 Thread via GitHub
cmccabe commented on code in PR #13345: URL: https://github.com/apache/kafka/pull/13345#discussion_r1146559697 ## raft/src/main/java/org/apache/kafka/raft/internals/ByteBufferSerde.java: ## @@ -0,0 +1,44 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one or

[GitHub] [kafka] cmccabe commented on a diff in pull request #13345: KAFKA-13020; Implement reading Snapshot log append timestamp

2023-03-23 Thread via GitHub
cmccabe commented on code in PR #13345: URL: https://github.com/apache/kafka/pull/13345#discussion_r1146542326 ## raft/src/main/java/org/apache/kafka/snapshot/RecordsSnapshotReader.java: ## @@ -121,9 +122,22 @@ private Optional> nextBatch() { Batch batch =

[GitHub] [kafka] cmccabe commented on a diff in pull request #13345: KAFKA-13020; Implement reading Snapshot log append timestamp

2023-03-23 Thread via GitHub
cmccabe commented on code in PR #13345: URL: https://github.com/apache/kafka/pull/13345#discussion_r1146533884 ## raft/src/main/java/org/apache/kafka/raft/Batch.java: ## @@ -33,21 +33,24 @@ private final int sizeInBytes; private final long lastOffset; private

[GitHub] [kafka] chia7712 commented on a diff in pull request #13393: KAFKA-10244 An new java interface to replace 'kafka.common.MessageReader'

2023-03-23 Thread via GitHub
chia7712 commented on code in PR #13393: URL: https://github.com/apache/kafka/pull/13393#discussion_r1146517888 ## core/src/main/scala/kafka/tools/ConsoleProducer.scala: ## @@ -24,33 +24,81 @@ import java.util.regex.Pattern import joptsimple.{OptionException, OptionParser,

[jira] [Assigned] (KAFKA-14420) MirrorMaker should not clear filtered configs on target topics

2023-03-23 Thread Mickael Maison (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-14420?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Mickael Maison reassigned KAFKA-14420: -- Assignee: Gantigmaa Selenge (was: Mickael Maison) > MirrorMaker should not clear

[jira] [Created] (KAFKA-14839) Exclude protected variable from JavaDocs

2023-03-23 Thread Matthias J. Sax (Jira)
Matthias J. Sax created KAFKA-14839: --- Summary: Exclude protected variable from JavaDocs Key: KAFKA-14839 URL: https://issues.apache.org/jira/browse/KAFKA-14839 Project: Kafka Issue Type:

[GitHub] [kafka] mimaison commented on a diff in pull request #13393: KAFKA-10244 An new java interface to replace 'kafka.common.MessageReader'

2023-03-23 Thread via GitHub
mimaison commented on code in PR #13393: URL: https://github.com/apache/kafka/pull/13393#discussion_r1146437275 ## clients/src/main/java/org/apache/kafka/tools/RecordReader.java: ## @@ -0,0 +1,51 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one or more + *

[GitHub] [kafka] guozhangwang merged pull request #13427: MINOR:Incorrect/canonical use of constants in AdminClientConfig and StreamsConfigTest

2023-03-23 Thread via GitHub
guozhangwang merged PR #13427: URL: https://github.com/apache/kafka/pull/13427 -- 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:

[GitHub] [kafka] guozhangwang commented on pull request #13427: MINOR:Incorrect/canonical use of constants in AdminClientConfig and StreamsConfigTest

2023-03-23 Thread via GitHub
guozhangwang commented on PR #13427: URL: https://github.com/apache/kafka/pull/13427#issuecomment-1481514325 Thanks for the catch! LGTM. -- 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

[GitHub] [kafka] cmccabe merged pull request #13439: MINOR: Rename remote_kraft to isolated_kraft in kafkatests

2023-03-23 Thread via GitHub
cmccabe merged PR #13439: URL: https://github.com/apache/kafka/pull/13439 -- 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] [Commented] (KAFKA-14837) The MirrorCheckPointConnector of MM2 will rebalance frequently, when the source cluster group is many more and changes frequently (but the list of configured synchrono

2023-03-23 Thread Chris Egerton (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-14837?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel=17704244#comment-17704244 ] Chris Egerton commented on KAFKA-14837: --- [~hudeqi] have you tried adjusting the

[GitHub] [kafka] mumrah commented on a diff in pull request #13438: KAFKA-14835: Create ControllerServerMetricsPublisher

2023-03-23 Thread via GitHub
mumrah commented on code in PR #13438: URL: https://github.com/apache/kafka/pull/13438#discussion_r1146431114 ## core/src/main/scala/kafka/server/ControllerServer.scala: ## @@ -366,6 +376,8 @@ class ControllerServer( // Ensure that we're not the Raft leader prior to

[GitHub] [kafka] C0urante commented on a diff in pull request #13434: KAFKA-14785: Connect offset read REST API

2023-03-23 Thread via GitHub
C0urante commented on code in PR #13434: URL: https://github.com/apache/kafka/pull/13434#discussion_r1146407254 ## connect/runtime/src/main/java/org/apache/kafka/connect/storage/OffsetBackingStore.java: ## @@ -65,6 +66,13 @@ public interface OffsetBackingStore { */

[GitHub] [kafka] C0urante commented on a diff in pull request #13434: KAFKA-14785: Connect offset read REST API

2023-03-23 Thread via GitHub
C0urante commented on code in PR #13434: URL: https://github.com/apache/kafka/pull/13434#discussion_r1146402215 ## connect/runtime/src/main/java/org/apache/kafka/connect/runtime/Worker.java: ## @@ -1133,6 +1139,106 @@ public void setTargetState(String connName, TargetState

[GitHub] [kafka] C0urante commented on a diff in pull request #13434: KAFKA-14785: Connect offset read REST API

2023-03-23 Thread via GitHub
C0urante commented on code in PR #13434: URL: https://github.com/apache/kafka/pull/13434#discussion_r1146391973 ## connect/runtime/src/main/java/org/apache/kafka/connect/runtime/Worker.java: ## @@ -1133,6 +1139,106 @@ public void setTargetState(String connName, TargetState

[GitHub] [kafka] C0urante commented on a diff in pull request #13434: KAFKA-14785: Connect offset read REST API

2023-03-23 Thread via GitHub
C0urante commented on code in PR #13434: URL: https://github.com/apache/kafka/pull/13434#discussion_r1146391973 ## connect/runtime/src/main/java/org/apache/kafka/connect/runtime/Worker.java: ## @@ -1133,6 +1139,106 @@ public void setTargetState(String connName, TargetState

[GitHub] [kafka] C0urante commented on a diff in pull request #13434: KAFKA-14785: Connect offset read REST API

2023-03-23 Thread via GitHub
C0urante commented on code in PR #13434: URL: https://github.com/apache/kafka/pull/13434#discussion_r1146391973 ## connect/runtime/src/main/java/org/apache/kafka/connect/runtime/Worker.java: ## @@ -1133,6 +1139,106 @@ public void setTargetState(String connName, TargetState

[jira] [Resolved] (KAFKA-14814) Skip restart of connectors when redundant resume request is made

2023-03-23 Thread Chris Egerton (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-14814?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Chris Egerton resolved KAFKA-14814. --- Fix Version/s: 3.5.0 Resolution: Fixed > Skip restart of connectors when redundant

[GitHub] [kafka] C0urante merged pull request #13426: KAFKA-14814: Skip target state updates when the configs store has sam…

2023-03-23 Thread via GitHub
C0urante merged PR #13426: URL: https://github.com/apache/kafka/pull/13426 -- 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:

[GitHub] [kafka] C0urante commented on a diff in pull request #13426: KAFKA-14814: Skip target state updates when the configs store has sam…

2023-03-23 Thread via GitHub
C0urante commented on code in PR #13426: URL: https://github.com/apache/kafka/pull/13426#discussion_r1146367160 ## connect/runtime/src/test/java/org/apache/kafka/connect/storage/KafkaConfigBackingStoreTest.java: ## @@ -889,6 +890,51 @@ public void

[jira] [Updated] (KAFKA-14652) Improve MM2 logging by adding the flow information to the context

2023-03-23 Thread Daniel Urban (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-14652?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Daniel Urban updated KAFKA-14652: - Description: MirrorMaker2 runs multiple Connect worker instances in a single process. In

[jira] [Created] (KAFKA-14838) MM2 Connector/Task clients should specify client ID based on ID and role

2023-03-23 Thread Daniel Urban (Jira)
Daniel Urban created KAFKA-14838: Summary: MM2 Connector/Task clients should specify client ID based on ID and role Key: KAFKA-14838 URL: https://issues.apache.org/jira/browse/KAFKA-14838 Project:

[jira] [Assigned] (KAFKA-14652) Improve MM2 logging by adding the flow information to the context

2023-03-23 Thread Daniel Urban (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-14652?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Daniel Urban reassigned KAFKA-14652: Assignee: Daniel Urban > Improve MM2 logging by adding the flow information to the

[GitHub] [kafka] C0urante commented on pull request #13424: KAFKA-14783 (KIP-875): New STOPPED state for connectors

2023-03-23 Thread via GitHub
C0urante commented on PR #13424: URL: https://github.com/apache/kafka/pull/13424#issuecomment-1481323296 Thanks all for the reviews! I believe I've addressed every comment with either a code change or a response. This should be ready for another pass. -- This is an automated message from

[GitHub] [kafka] C0urante commented on a diff in pull request #13424: KAFKA-14783 (KIP-875): New STOPPED state for connectors

2023-03-23 Thread via GitHub
C0urante commented on code in PR #13424: URL: https://github.com/apache/kafka/pull/13424#discussion_r1146280630 ## connect/runtime/src/main/java/org/apache/kafka/connect/runtime/standalone/StandaloneHerder.java: ## @@ -424,6 +435,9 @@ private void updateConnectorTasks(String

[GitHub] [kafka] C0urante commented on a diff in pull request #13424: KAFKA-14783 (KIP-875): New STOPPED state for connectors

2023-03-23 Thread via GitHub
C0urante commented on code in PR #13424: URL: https://github.com/apache/kafka/pull/13424#discussion_r1146278701 ## connect/runtime/src/main/java/org/apache/kafka/connect/runtime/distributed/DistributedHerder.java: ## @@ -1090,6 +1090,39 @@ public void putConnectorConfig(final

[GitHub] [kafka] C0urante commented on a diff in pull request #13424: KAFKA-14783 (KIP-875): New STOPPED state for connectors

2023-03-23 Thread via GitHub
C0urante commented on code in PR #13424: URL: https://github.com/apache/kafka/pull/13424#discussion_r1146278053 ## connect/runtime/src/test/java/org/apache/kafka/connect/util/clusters/EmbeddedConnectCluster.java: ## @@ -553,6 +570,54 @@ public ActiveTopicsInfo

[GitHub] [kafka] C0urante commented on a diff in pull request #13424: KAFKA-14783 (KIP-875): New STOPPED state for connectors

2023-03-23 Thread via GitHub
C0urante commented on code in PR #13424: URL: https://github.com/apache/kafka/pull/13424#discussion_r1146277661 ## connect/runtime/src/main/java/org/apache/kafka/connect/runtime/rest/resources/ConnectorsResource.java: ## @@ -274,6 +274,19 @@ public Response

[jira] [Assigned] (KAFKA-14837) The MirrorCheckPointConnector of MM2 will rebalance frequently, when the source cluster group is many more and changes frequently (but the list of configured synchronou

2023-03-23 Thread hudeqi (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-14837?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] hudeqi reassigned KAFKA-14837: -- Assignee: hudeqi > The MirrorCheckPointConnector of MM2 will rebalance frequently, when the >

[GitHub] [kafka] C0urante commented on a diff in pull request #13424: KAFKA-14783 (KIP-875): New STOPPED state for connectors

2023-03-23 Thread via GitHub
C0urante commented on code in PR #13424: URL: https://github.com/apache/kafka/pull/13424#discussion_r1146277240 ## connect/runtime/src/test/java/org/apache/kafka/connect/integration/ConnectWorkerIntegrationTest.java: ## @@ -325,6 +325,181 @@ public void

[GitHub] [kafka] C0urante commented on a diff in pull request #13424: KAFKA-14783 (KIP-875): New STOPPED state for connectors

2023-03-23 Thread via GitHub
C0urante commented on code in PR #13424: URL: https://github.com/apache/kafka/pull/13424#discussion_r1146276968 ## connect/runtime/src/test/java/org/apache/kafka/connect/util/clusters/EmbeddedConnectClusterAssertions.java: ## @@ -450,6 +480,35 @@ protected boolean

[GitHub] [kafka] C0urante commented on a diff in pull request #13424: KAFKA-14783 (KIP-875): New STOPPED state for connectors

2023-03-23 Thread via GitHub
C0urante commented on code in PR #13424: URL: https://github.com/apache/kafka/pull/13424#discussion_r1146276523 ## connect/runtime/src/main/java/org/apache/kafka/connect/runtime/WorkerConnector.java: ## @@ -221,28 +223,44 @@ public boolean isRunning() { }

[GitHub] [kafka] C0urante commented on a diff in pull request #13424: KAFKA-14783 (KIP-875): New STOPPED state for connectors

2023-03-23 Thread via GitHub
C0urante commented on code in PR #13424: URL: https://github.com/apache/kafka/pull/13424#discussion_r1146276098 ## connect/runtime/src/main/java/org/apache/kafka/connect/runtime/Herder.java: ## @@ -245,6 +245,14 @@ default void validateConnectorConfig(Map connectorConfig,

[jira] [Updated] (KAFKA-14837) The MirrorCheckPointConnector of MM2 will rebalance frequently, when the source cluster group is many more and changes frequently (but the list of configured synchronous

2023-03-23 Thread hudeqi (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-14837?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] hudeqi updated KAFKA-14837: --- Description: In practice, I found that when I configure a mirror checkpoint connector, because the source

[jira] [Created] (KAFKA-14837) The MirrorCheckPointConnector of MM2 will rebalance frequently, when the source cluster group is many more and changes frequently (but the list of configured synchronous

2023-03-23 Thread hudeqi (Jira)
hudeqi created KAFKA-14837: -- Summary: The MirrorCheckPointConnector of MM2 will rebalance frequently, when the source cluster group is many more and changes frequently (but the list of configured synchronous group does not change) Key:

[GitHub] [kafka] C0urante commented on a diff in pull request #13424: KAFKA-14783 (KIP-875): New STOPPED state for connectors

2023-03-23 Thread via GitHub
C0urante commented on code in PR #13424: URL: https://github.com/apache/kafka/pull/13424#discussion_r1146250479 ## connect/runtime/src/test/java/org/apache/kafka/connect/integration/ConnectWorkerIntegrationTest.java: ## @@ -325,6 +325,181 @@ public void

[jira] [Updated] (KAFKA-14836) Fix UtilsTest#testToLogDateTimeFormat failure in some cases

2023-03-23 Thread Tamas Barnabas Egyed (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-14836?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Tamas Barnabas Egyed updated KAFKA-14836: - Summary: Fix UtilsTest#testToLogDateTimeFormat failure in some cases (was:

[GitHub] [kafka] yashmayya commented on a diff in pull request #13424: KAFKA-14783 (KIP-875): New STOPPED state for connectors

2023-03-23 Thread via GitHub
yashmayya commented on code in PR #13424: URL: https://github.com/apache/kafka/pull/13424#discussion_r1146151628 ## connect/runtime/src/test/java/org/apache/kafka/connect/runtime/distributed/DistributedHerderTest.java: ## @@ -215,6 +215,17 @@ public class DistributedHerderTest

[jira] [Created] (KAFKA-14836) UtilsTest testToLogDateTimeFormat unit test failing in some cases

2023-03-23 Thread Tamas Barnabas Egyed (Jira)
Tamas Barnabas Egyed created KAFKA-14836: Summary: UtilsTest testToLogDateTimeFormat unit test failing in some cases Key: KAFKA-14836 URL: https://issues.apache.org/jira/browse/KAFKA-14836

[GitHub] [kafka] andymg3 opened a new pull request, #13440: KAFKA-14829: Consolidate reassignment logic in PartitionReassignmentReplicas

2023-03-23 Thread via GitHub
andymg3 opened a new pull request, #13440: URL: https://github.com/apache/kafka/pull/13440 ### JIRA https://issues.apache.org/jira/browse/KAFKA-14829 ### Details Currently, we have various bits of reassignment logic spread across different classes. For example,

[GitHub] [kafka] mukkachaitanya commented on pull request #13361: KAFKA-14401: Resume WorkThread if Connector/Tasks reading offsets get stuck when underneath WorkThread dies

2023-03-23 Thread via GitHub
mukkachaitanya commented on PR #13361: URL: https://github.com/apache/kafka/pull/13361#issuecomment-1480988122 > I don't think that we should add retries when we already know that the exceptions that would be caught here are non-retriable. Additionally, it may be unsafe or incorrect to

[GitHub] [kafka] yashmayya commented on a diff in pull request #13424: KAFKA-14783 (KIP-875): New STOPPED state for connectors

2023-03-23 Thread via GitHub
yashmayya commented on code in PR #13424: URL: https://github.com/apache/kafka/pull/13424#discussion_r1144717494 ## connect/runtime/src/test/java/org/apache/kafka/connect/integration/ConnectWorkerIntegrationTest.java: ## @@ -325,6 +325,181 @@ public void

  1   2   >