Re: [PR] Ctr end offsets shoud not throw [kafka]

2024-03-12 Thread via GitHub
PoliakovArtur commented on code in PR #15525: URL: https://github.com/apache/kafka/pull/15525#discussion_r1522610604 ## clients/src/main/java/org/apache/kafka/clients/consumer/internals/OffsetFetcher.java: ## @@ -398,6 +401,7 @@ private RequestFuture sendListOffsetRequest(final

Re: [PR] Ctr end offsets shoud not throw [kafka]

2024-03-12 Thread via GitHub
PoliakovArtur commented on code in PR #15525: URL: https://github.com/apache/kafka/pull/15525#discussion_r1522610604 ## clients/src/main/java/org/apache/kafka/clients/consumer/internals/OffsetFetcher.java: ## @@ -398,6 +401,7 @@ private RequestFuture sendListOffsetRequest(final

Re: [PR] MINOR: Remove deprecation and exception throw in ProcessorRecordContext#hashcode [kafka]

2024-03-12 Thread via GitHub
ableegoldman commented on PR #15508: URL: https://github.com/apache/kafka/pull/15508#issuecomment-1993664255 My official take on this is: `Headers` is a stupid class. Why is it mutable?? Seems very silly to me Though I'm sure there is (or at least _was_) a reason... Anyways, pu

Re: [PR] MINOR: Remove deprecation and exception throw in ProcessorRecordContext#hashcode [kafka]

2024-03-12 Thread via GitHub
ableegoldman commented on PR #15508: URL: https://github.com/apache/kafka/pull/15508#issuecomment-1993658568 Ah, thanks for the reality check guys -- I always forget about `Headers` being mutable 😞 Glad John deprecated the method way back when since the deprecation warning was how I

Re: [PR] KAFKA-16314: Introducing the AbortableTransactionException [kafka]

2024-03-12 Thread via GitHub
sjhajharia commented on code in PR #15486: URL: https://github.com/apache/kafka/pull/15486#discussion_r1522533304 ## core/src/main/scala/kafka/server/AddPartitionsToTxnManager.scala: ## @@ -42,6 +42,11 @@ object AddPartitionsToTxnManager { val VerificationTimeMsMetricName = "

Re: [PR] KAFKA-16341: fix the LogValidator for non-compressed type [kafka]

2024-03-12 Thread via GitHub
showuon commented on code in PR #15476: URL: https://github.com/apache/kafka/pull/15476#discussion_r1522451780 ## storage/src/main/java/org/apache/kafka/storage/internals/log/LogValidator.java: ## @@ -293,11 +293,11 @@ public ValidationResult assignOffsetsNonCompressed(LongRef

Re: [PR] [DO NOT MERGE] KAFKA-14419: limit time spent processing during ongoing rebalance and delay followup rebalance trigger [kafka]

2024-03-12 Thread via GitHub
github-actions[bot] commented on PR #15009: URL: https://github.com/apache/kafka/pull/15009#issuecomment-1993330968 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 merge

Re: [PR] KAFKA-16342: fix getOffsetByMaxTimestamp for compressed records [kafka]

2024-03-12 Thread via GitHub
showuon commented on code in PR #15474: URL: https://github.com/apache/kafka/pull/15474#discussion_r1522447288 ## clients/src/main/java/org/apache/kafka/common/record/MemoryRecordsBuilder.java: ## @@ -242,34 +242,23 @@ public MemoryRecords build() { /** * Get the ma

Re: [PR] KAFKA-16342: fix getOffsetByMaxTimestamp for compressed records [kafka]

2024-03-12 Thread via GitHub
showuon commented on code in PR #15474: URL: https://github.com/apache/kafka/pull/15474#discussion_r1522447288 ## clients/src/main/java/org/apache/kafka/common/record/MemoryRecordsBuilder.java: ## @@ -242,34 +242,23 @@ public MemoryRecords build() { /** * Get the ma

Re: [PR] KAFKA-16342: fix getOffsetByMaxTimestamp for compressed records [kafka]

2024-03-12 Thread via GitHub
showuon commented on code in PR #15474: URL: https://github.com/apache/kafka/pull/15474#discussion_r1522447288 ## clients/src/main/java/org/apache/kafka/common/record/MemoryRecordsBuilder.java: ## @@ -242,34 +242,23 @@ public MemoryRecords build() { /** * Get the ma

Re: [PR] KAFKA-15561: Client support for new SubscriptionPattern based subscription [kafka]

2024-03-12 Thread via GitHub
Phuc-Hong-Tran commented on PR #15188: URL: https://github.com/apache/kafka/pull/15188#issuecomment-1993206506 Regex validity check will be included in the next pull request, I'll try to get it done by this weekend. -- This is an automated message from the Apache Git Service. To respond t

Re: [PR] KAFKA-15716: KRaft support in EpochDrivenReplicationProtocolAcceptanceTest [kafka]

2024-03-12 Thread via GitHub
highluck commented on code in PR #15295: URL: https://github.com/apache/kafka/pull/15295#discussion_r1522384690 ## core/src/test/scala/unit/kafka/server/epoch/EpochDrivenReplicationProtocolAcceptanceTest.scala: ## @@ -53,17 +58,26 @@ class EpochDrivenReplicationProtocolAcceptanc

Re: [PR] KAFKA-15716: KRaft support in EpochDrivenReplicationProtocolAcceptanceTest [kafka]

2024-03-12 Thread via GitHub
highluck commented on PR #15295: URL: https://github.com/apache/kafka/pull/15295#issuecomment-1993132575 @mimaison thanks for review I felt like I needed to think a little more, so I thought it would be a good idea to work on a follow-up PR. This is because the test does not wor

Re: [PR] KAFKA-16249; Improve reconciliation state machine [kafka]

2024-03-12 Thread via GitHub
jolshan commented on code in PR #15364: URL: https://github.com/apache/kafka/pull/15364#discussion_r1522334545 ## group-coordinator/src/main/java/org/apache/kafka/coordinator/group/GroupMetadataManager.java: ## @@ -1211,13 +1192,99 @@ private CoordinatorResult consumerGr

Re: [PR] KAFKA-16249; Improve reconciliation state machine [kafka]

2024-03-12 Thread via GitHub
jolshan commented on code in PR #15364: URL: https://github.com/apache/kafka/pull/15364#discussion_r1522334545 ## group-coordinator/src/main/java/org/apache/kafka/coordinator/group/GroupMetadataManager.java: ## @@ -1211,13 +1192,99 @@ private CoordinatorResult consumerGr

[jira] [Commented] (KAFKA-16354) FinalizedFeatureChangeListenerTest should use mocked latches

2024-03-12 Thread PoAn Yang (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-16354?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17825871#comment-17825871 ] PoAn Yang commented on KAFKA-16354: --- Hi [~gharris1727], it looks like not both `testC

Re: [PR] KAFKA-16341: fix the LogValidator for non-compressed type [kafka]

2024-03-12 Thread via GitHub
chia7712 commented on code in PR #15476: URL: https://github.com/apache/kafka/pull/15476#discussion_r1522318718 ## storage/src/main/java/org/apache/kafka/storage/internals/log/LogValidator.java: ## @@ -293,11 +293,11 @@ public ValidationResult assignOffsetsNonCompressed(LongRef

Re: [PR] KAFKA-16249; Improve reconciliation state machine [kafka]

2024-03-12 Thread via GitHub
jolshan commented on code in PR #15364: URL: https://github.com/apache/kafka/pull/15364#discussion_r1522316891 ## group-coordinator/src/main/java/org/apache/kafka/coordinator/group/consumer/MemberState.java: ## @@ -0,0 +1,76 @@ +/* + * Licensed to the Apache Software Foundation

Re: [PR] MINOR; Make string from array [kafka]

2024-03-12 Thread via GitHub
jolshan commented on code in PR #15526: URL: https://github.com/apache/kafka/pull/15526#discussion_r1522315310 ## core/src/main/scala/kafka/log/LogManager.scala: ## @@ -1564,7 +1564,7 @@ object LogManager { Option(newTopicsImage.getPartition(topicId, partitionId)) match {

Re: [PR] MINOR: simplify consumer logic [kafka]

2024-03-12 Thread via GitHub
mjsax commented on PR #15519: URL: https://github.com/apache/kafka/pull/15519#issuecomment-1992854817 PR update does not show up right now. Github Status page: > Update - We're continuing to investigate an elevated number of pull requests that are out of sync on page load. Mar 13, 2024

Re: [PR] MINOR; Make string from array [kafka]

2024-03-12 Thread via GitHub
jsancio commented on code in PR #15526: URL: https://github.com/apache/kafka/pull/15526#discussion_r1522311967 ## core/src/main/scala/kafka/log/LogManager.scala: ## @@ -1564,7 +1564,7 @@ object LogManager { Option(newTopicsImage.getPartition(topicId, partitionId)) match {

Re: [PR] MINOR: simplify consumer logic [kafka]

2024-03-12 Thread via GitHub
mjsax commented on PR #15519: URL: https://github.com/apache/kafka/pull/15519#issuecomment-1992841168 Thank for all the input! -- Updated the PR accordinly. -- 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

Re: [PR] KAFKA-15417 Sloppy bug: moved the outerJounBreak-flags out of the loop [kafka]

2024-03-12 Thread via GitHub
mjsax commented on code in PR #15510: URL: https://github.com/apache/kafka/pull/15510#discussion_r1522304963 ## streams/src/main/java/org/apache/kafka/streams/kstream/internals/KStreamKStreamJoin.java: ## @@ -223,9 +223,9 @@ private void emitNonJoinedOuterRecords( t

Re: [PR] KAFKA-15417 Sloppy bug: moved the outerJounBreak-flags out of the loop [kafka]

2024-03-12 Thread via GitHub
mjsax commented on code in PR #15510: URL: https://github.com/apache/kafka/pull/15510#discussion_r1522304963 ## streams/src/main/java/org/apache/kafka/streams/kstream/internals/KStreamKStreamJoin.java: ## @@ -223,9 +223,9 @@ private void emitNonJoinedOuterRecords( t

Re: [PR] MINOR; Make string from array [kafka]

2024-03-12 Thread via GitHub
jolshan commented on code in PR #15526: URL: https://github.com/apache/kafka/pull/15526#discussion_r1522294411 ## core/src/main/scala/kafka/log/LogManager.scala: ## @@ -1564,7 +1564,7 @@ object LogManager { Option(newTopicsImage.getPartition(topicId, partitionId)) match {

Re: [PR] KAFKA-16342: fix getOffsetByMaxTimestamp for compressed records [kafka]

2024-03-12 Thread via GitHub
showuon commented on code in PR #15474: URL: https://github.com/apache/kafka/pull/15474#discussion_r1522294617 ## core/src/test/scala/integration/kafka/admin/ListOffsetsIntegrationTest.scala: ## @@ -19,82 +19,230 @@ package kafka.admin import kafka.integration.KafkaServerTest

Re: [PR] MINOR; Make string from array [kafka]

2024-03-12 Thread via GitHub
jolshan commented on code in PR #15526: URL: https://github.com/apache/kafka/pull/15526#discussion_r1522294411 ## core/src/main/scala/kafka/log/LogManager.scala: ## @@ -1564,7 +1564,7 @@ object LogManager { Option(newTopicsImage.getPartition(topicId, partitionId)) match {

Re: [PR] MINOR; Make string from array [kafka]

2024-03-12 Thread via GitHub
jolshan commented on code in PR #15526: URL: https://github.com/apache/kafka/pull/15526#discussion_r1522293491 ## core/src/main/scala/kafka/log/LogManager.scala: ## @@ -1564,7 +1564,7 @@ object LogManager { Option(newTopicsImage.getPartition(topicId, partitionId)) match {

[PR] MINOR; Make string from array [kafka]

2024-03-12 Thread via GitHub
jsancio opened a new pull request, #15526: URL: https://github.com/apache/kafka/pull/15526 If toString is called on an array it returns the string representing the object reference. Use mkString instead to print the content of the array. ### Committer Checklist (excluded from commit

Re: [PR] KAFKA-14683 Cleanup WorkerSinkTaskTest [kafka]

2024-03-12 Thread via GitHub
hgeraldino commented on PR #15506: URL: https://github.com/apache/kafka/pull/15506#issuecomment-1992692291 Thanks @OmniaGM for your review -- 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 spec

Re: [PR] KAFKA-16341: fix the LogValidator for non-compressed type [kafka]

2024-03-12 Thread via GitHub
junrao commented on code in PR #15476: URL: https://github.com/apache/kafka/pull/15476#discussion_r1522196043 ## storage/src/main/java/org/apache/kafka/storage/internals/log/LogValidator.java: ## @@ -293,11 +293,11 @@ public ValidationResult assignOffsetsNonCompressed(LongRef o

Re: [PR] KAFKA-14683 Cleanup WorkerSinkTaskTest [kafka]

2024-03-12 Thread via GitHub
hgeraldino commented on code in PR #15506: URL: https://github.com/apache/kafka/pull/15506#discussion_r1522194636 ## connect/runtime/src/test/java/org/apache/kafka/connect/runtime/WorkerSinkTaskTest.java: ## @@ -1215,17 +1195,13 @@ public void testSuppressCloseErrors() {

Re: [PR] MINOR: Tweak streams config doc [kafka]

2024-03-12 Thread via GitHub
mjsax merged PR #15518: URL: https://github.com/apache/kafka/pull/15518 -- 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-unsubscr...@kafka.apache.or

Re: [PR] KAFKA-16123: KStreamKStreamJoinProcessor does not drop late records. [kafka]

2024-03-12 Thread via GitHub
florin-akermann commented on code in PR #15189: URL: https://github.com/apache/kafka/pull/15189#discussion_r1522184075 ## streams/src/test/java/org/apache/kafka/streams/kstream/internals/KStreamKStreamWindowCloseTest.java: ## @@ -0,0 +1,150 @@ +/* + * Licensed to the Apache Soft

Re: [PR] KAFKA-16123: KStreamKStreamJoinProcessor does not drop late records. [kafka]

2024-03-12 Thread via GitHub
florin-akermann commented on code in PR #15189: URL: https://github.com/apache/kafka/pull/15189#discussion_r1522184075 ## streams/src/test/java/org/apache/kafka/streams/kstream/internals/KStreamKStreamWindowCloseTest.java: ## @@ -0,0 +1,150 @@ +/* + * Licensed to the Apache Soft

Re: [PR] KAFKA-16342: fix getOffsetByMaxTimestamp for compressed records [kafka]

2024-03-12 Thread via GitHub
junrao commented on code in PR #15474: URL: https://github.com/apache/kafka/pull/15474#discussion_r1522164475 ## core/src/test/scala/integration/kafka/admin/ListOffsetsIntegrationTest.scala: ## @@ -19,82 +19,230 @@ package kafka.admin import kafka.integration.KafkaServerTestH

Re: [PR] KAFKA-14589 ConsumerGroupCommand rewritten in java [kafka]

2024-03-12 Thread via GitHub
nizhikov commented on PR #14471: URL: https://github.com/apache/kafka/pull/14471#issuecomment-1992664069 @chia7712 I ran system tests for command, also. System tests passed. ``` SESSION REPOR

Re: [PR] KAFKA-14589 ConsumerGroupCommand rewritten in java [kafka]

2024-03-12 Thread via GitHub
nizhikov commented on code in PR #14471: URL: https://github.com/apache/kafka/pull/14471#discussion_r1522059985 ## build.gradle: ## @@ -1999,10 +1999,14 @@ project(':tools') { implementation project(':clients') implementation project(':storage') implementation pro

[jira] [Commented] (KAFKA-16359) kafka-clients-3.7.0.jar published to Maven Central is defective

2024-03-12 Thread Ismael Juma (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-16359?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17825794#comment-17825794 ] Ismael Juma commented on KAFKA-16359: - [~norrisjeremy] it's clearly a bug - there is

Re: [PR] KAFKA-16342: fix getOffsetByMaxTimestamp for compressed records [kafka]

2024-03-12 Thread via GitHub
chia7712 commented on code in PR #15474: URL: https://github.com/apache/kafka/pull/15474#discussion_r1522051894 ## clients/src/main/java/org/apache/kafka/common/record/MemoryRecordsBuilder.java: ## @@ -243,33 +243,20 @@ public MemoryRecords build() { /** * Get the max

[PR] Ctr end offsets shoud not throw [kafka]

2024-03-12 Thread via GitHub
philipnee opened a new pull request, #15525: URL: https://github.com/apache/kafka/pull/15525 *More detailed description of your change, if necessary. The PR title and PR message become the squashed commit message, so use a separate comment to ping reviewers.* *Summary of testi

[jira] [Commented] (KAFKA-16359) kafka-clients-3.7.0.jar published to Maven Central is defective

2024-03-12 Thread Apoorv Mittal (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-16359?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17825772#comment-17825772 ] Apoorv Mittal commented on KAFKA-16359: --- Though by the time fix and release happen

[jira] [Commented] (KAFKA-16359) kafka-clients-3.7.0.jar published to Maven Central is defective

2024-03-12 Thread Jeremy Norris (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-16359?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17825770#comment-17825770 ] Jeremy Norris commented on KAFKA-16359: --- It's broken because users that attempt to

[jira] [Commented] (KAFKA-16359) kafka-clients-3.7.0.jar published to Maven Central is defective

2024-03-12 Thread Ismael Juma (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-16359?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17825767#comment-17825767 ] Ismael Juma commented on KAFKA-16359: - The original message said there were warnings

Re: [PR] KAFKA-16314: Introducing the AbortableTransactionException [kafka]

2024-03-12 Thread via GitHub
jolshan commented on code in PR #15486: URL: https://github.com/apache/kafka/pull/15486#discussion_r1521977816 ## core/src/main/scala/kafka/server/AddPartitionsToTxnManager.scala: ## @@ -42,6 +42,11 @@ object AddPartitionsToTxnManager { val VerificationTimeMsMetricName = "Ver

Re: [PR] KAFKA-16314: Introducing the AbortableTransactionException [kafka]

2024-03-12 Thread via GitHub
jolshan commented on code in PR #15486: URL: https://github.com/apache/kafka/pull/15486#discussion_r1521974720 ## core/src/main/scala/kafka/server/AddPartitionsToTxnManager.scala: ## @@ -42,6 +42,11 @@ object AddPartitionsToTxnManager { val VerificationTimeMsMetricName = "Ver

[jira] [Commented] (KAFKA-16352) Transaction may get get stuck in PrepareCommit or PrepareAbort state

2024-03-12 Thread Artem Livshits (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-16352?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17825766#comment-17825766 ] Artem Livshits commented on KAFKA-16352: that's correct: the transaction is full

Re: [PR] KAFKA-16352: Txn may get get stuck in PrepareCommit or PrepareAbort s… [kafka]

2024-03-12 Thread via GitHub
artemlivshits commented on code in PR #15524: URL: https://github.com/apache/kafka/pull/15524#discussion_r1521968243 ## core/src/main/scala/kafka/coordinator/transaction/TransactionMarkerChannelManager.scala: ## @@ -354,41 +366,42 @@ class TransactionMarkerChannelManager(

Re: [PR] KAFKA-16318 : add javafoc for kafka metric [kafka]

2024-03-12 Thread via GitHub
chia7712 commented on code in PR #15483: URL: https://github.com/apache/kafka/pull/15483#discussion_r1521949713 ## clients/src/main/java/org/apache/kafka/common/metrics/KafkaMetric.java: ## @@ -40,15 +48,28 @@ public KafkaMetric(Object lock, MetricName metricName, MetricValuePr

[jira] [Resolved] (KAFKA-9690) MemoryLeak in JMX Reporter

2024-03-12 Thread Chia-Ping Tsai (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-9690?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Chia-Ping Tsai resolved KAFKA-9690. --- Resolution: Duplicate close this due to https://issues.apache.org/jira/browse/KAFKA-9306 > M

[jira] [Resolved] (KAFKA-9504) Memory leak in KafkaMetrics registered to MBean

2024-03-12 Thread Chia-Ping Tsai (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-9504?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Chia-Ping Tsai resolved KAFKA-9504. --- Resolution: Duplicate close this due to https://issues.apache.org/jira/browse/KAFKA-9306 > M

[PR] KAFKA-16352: Txn may get get stuck in PrepareCommit or PrepareAbort s… [kafka]

2024-03-12 Thread via GitHub
artemlivshits opened a new pull request, #15524: URL: https://github.com/apache/kafka/pull/15524 …tate Now the removal of entries from the transactionsWithPendingMarkers map checks the value and all pending marker operations keep the value along with the operation state. This way, t

Re: [PR] KAFKA-16312, KAFKA-16185: Local epochs in reconciliation [kafka]

2024-03-12 Thread via GitHub
lianetm commented on code in PR #15511: URL: https://github.com/apache/kafka/pull/15511#discussion_r1521937262 ## clients/src/main/java/org/apache/kafka/clients/consumer/internals/MembershipManagerImpl.java: ## @@ -210,12 +213,12 @@ public class MembershipManagerImpl implements

Re: [PR] KAFKA-14589 ConsumerGroupCommand rewritten in java [kafka]

2024-03-12 Thread via GitHub
chia7712 commented on code in PR #14471: URL: https://github.com/apache/kafka/pull/14471#discussion_r1521936219 ## build.gradle: ## @@ -1999,10 +1999,14 @@ project(':tools') { implementation project(':clients') implementation project(':storage') implementation pro

Re: [PR] KAFKA-16312, KAFKA-16185: Local epochs in reconciliation [kafka]

2024-03-12 Thread via GitHub
lianetm commented on code in PR #15511: URL: https://github.com/apache/kafka/pull/15511#discussion_r1521933499 ## clients/src/main/java/org/apache/kafka/clients/consumer/internals/MembershipManagerImpl.java: ## @@ -889,43 +914,36 @@ private void transitionToStale() { */

Re: [PR] KAFKA-16312, KAFKA-16185: Local epochs in reconciliation [kafka]

2024-03-12 Thread via GitHub
lianetm commented on code in PR #15511: URL: https://github.com/apache/kafka/pull/15511#discussion_r1521933499 ## clients/src/main/java/org/apache/kafka/clients/consumer/internals/MembershipManagerImpl.java: ## @@ -889,43 +914,36 @@ private void transitionToStale() { */

Re: [PR] KAFKA-15417 Sloppy bug: moved the outerJounBreak-flags out of the loop [kafka]

2024-03-12 Thread via GitHub
VictorvandenHoven commented on code in PR #15510: URL: https://github.com/apache/kafka/pull/15510#discussion_r1521827156 ## streams/src/main/java/org/apache/kafka/streams/kstream/internals/KStreamKStreamJoin.java: ## @@ -223,9 +223,9 @@ private void emitNonJoinedOuterRecords(

[jira] [Updated] (KAFKA-15250) ConsumerNetworkThread is running tight loop

2024-03-12 Thread Kirk True (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-15250?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Kirk True updated KAFKA-15250: -- Summary: ConsumerNetworkThread is running tight loop (was: DefaultBackgroundThread is running tight l

Re: [PR] KAFKA-16314: Introducing the AbortableTransactionException [kafka]

2024-03-12 Thread via GitHub
jolshan commented on code in PR #15486: URL: https://github.com/apache/kafka/pull/15486#discussion_r1521780875 ## core/src/main/scala/kafka/server/ReplicaManager.scala: ## @@ -1007,7 +1010,8 @@ class ReplicaManager(val config: KafkaConfig, transactionalId, producer

Re: [PR] KAFKA-16314: Introducing the AbortableTransactionException [kafka]

2024-03-12 Thread via GitHub
sjhajharia commented on code in PR #15486: URL: https://github.com/apache/kafka/pull/15486#discussion_r1521767609 ## core/src/main/scala/kafka/server/ReplicaManager.scala: ## @@ -1007,7 +1010,8 @@ class ReplicaManager(val config: KafkaConfig, transactionalId, produ

Re: [PR] KAFKA-16312, KAFKA-16185: Local epochs in reconciliation [kafka]

2024-03-12 Thread via GitHub
lucasbru commented on code in PR #15511: URL: https://github.com/apache/kafka/pull/15511#discussion_r1521750747 ## clients/src/main/java/org/apache/kafka/clients/consumer/internals/MembershipManagerImpl.java: ## @@ -508,9 +508,30 @@ private void processAssignmentReceived(Consum

Re: [PR] KAFKA-16312, KAFKA-16185: Local epochs in reconciliation [kafka]

2024-03-12 Thread via GitHub
lianetm commented on code in PR #15511: URL: https://github.com/apache/kafka/pull/15511#discussion_r1521745679 ## clients/src/main/java/org/apache/kafka/clients/consumer/internals/MembershipManagerImpl.java: ## @@ -508,9 +508,30 @@ private void processAssignmentReceived(Consume

Re: [PR] MINOR: Cleanup BoundedList to Make Constructors More Safe [kafka]

2024-03-12 Thread via GitHub
ChrisAHolland commented on code in PR #15507: URL: https://github.com/apache/kafka/pull/15507#discussion_r1521727181 ## server-common/src/main/java/org/apache/kafka/server/mutable/BoundedList.java: ## @@ -35,24 +35,12 @@ public class BoundedList implements List { private fi

Re: [PR] MINOR: Cleanup BoundedList to Make Constructors More Safe [kafka]

2024-03-12 Thread via GitHub
ChrisAHolland commented on code in PR #15507: URL: https://github.com/apache/kafka/pull/15507#discussion_r1521713491 ## server-common/src/main/java/org/apache/kafka/server/mutable/BoundedList.java: ## @@ -35,24 +35,12 @@ public class BoundedList implements List { private fi

Re: [PR] KAFKA-16314: Introducing the AbortableTransactionException [kafka]

2024-03-12 Thread via GitHub
jolshan commented on code in PR #15486: URL: https://github.com/apache/kafka/pull/15486#discussion_r1521679508 ## core/src/main/scala/kafka/server/ReplicaManager.scala: ## @@ -1007,7 +1010,8 @@ class ReplicaManager(val config: KafkaConfig, transactionalId, producer

Re: [PR] KAFKA-15206: Fix the flaky RemoteIndexCacheTest.testClose test [kafka]

2024-03-12 Thread via GitHub
kamalcph commented on PR #15523: URL: https://github.com/apache/kafka/pull/15523#issuecomment-1991918079 > Thanks for the change, @kamalcph! Just for my knowledge, why is there a possibility that the Shutdownable thread never ran in the CI for this test? Due to resource constraint, `S

Re: [PR] KAFKA-15949: Unify metadata.version format in log and error message [kafka]

2024-03-12 Thread via GitHub
FrankYang0529 commented on code in PR #15505: URL: https://github.com/apache/kafka/pull/15505#discussion_r1521640919 ## core/src/main/scala/kafka/tools/StorageTool.scala: ## @@ -139,7 +139,7 @@ object StorageTool extends Logging { action(storeTrue()) formatParser.add

Re: [PR] KAFKA-16365: AssignmentsManager callback handling issues [kafka]

2024-03-12 Thread via GitHub
soarez commented on code in PR #15521: URL: https://github.com/apache/kafka/pull/15521#discussion_r1521634948 ## server/src/main/java/org/apache/kafka/server/AssignmentsManager.java: ## @@ -202,9 +199,12 @@ public void run() throws Exception { return;

Re: [PR] KAFKA-16312, KAFKA-16185: Local epochs in reconciliation [kafka]

2024-03-12 Thread via GitHub
lianetm commented on code in PR #15511: URL: https://github.com/apache/kafka/pull/15511#discussion_r1521632809 ## clients/src/main/java/org/apache/kafka/clients/consumer/internals/HeartbeatRequestManager.java: ## @@ -538,16 +539,16 @@ public ConsumerGroupHeartbeatRequestData bu

Re: [PR] KAFKA-16365: AssignmentsManager callback handling issues [kafka]

2024-03-12 Thread via GitHub
gaurav-narula commented on code in PR #15521: URL: https://github.com/apache/kafka/pull/15521#discussion_r1521612209 ## server/src/main/java/org/apache/kafka/server/AssignmentsManager.java: ## @@ -202,9 +199,12 @@ public void run() throws Exception { return;

Re: [PR] KAFKA-16312, KAFKA-16185: Local epochs in reconciliation [kafka]

2024-03-12 Thread via GitHub
dajac commented on code in PR #15511: URL: https://github.com/apache/kafka/pull/15511#discussion_r1521552751 ## clients/src/main/java/org/apache/kafka/clients/consumer/internals/HeartbeatRequestManager.java: ## @@ -538,16 +539,16 @@ public ConsumerGroupHeartbeatRequestData buil

Re: [PR] KAFKA-16249; Improve reconciliation state machine [kafka]

2024-03-12 Thread via GitHub
dajac commented on PR #15364: URL: https://github.com/apache/kafka/pull/15364#issuecomment-1991716363 @jolshan Thanks for your comments. I addressed them. -- 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 t

Re: [PR] KAFKA-16249; Improve reconciliation state machine [kafka]

2024-03-12 Thread via GitHub
dajac commented on PR #15364: URL: https://github.com/apache/kafka/pull/15364#issuecomment-1991715637 > > This is a non-backward compatible change. I think that we should do this change to cleanup the record. As KIP-848 is only in early access in 3.7 and that we clearly state that we don't

Re: [PR] KAFKA-14683 Cleanup WorkerSinkTaskTest [kafka]

2024-03-12 Thread via GitHub
OmniaGM commented on code in PR #15506: URL: https://github.com/apache/kafka/pull/15506#discussion_r1521506327 ## connect/runtime/src/test/java/org/apache/kafka/connect/runtime/WorkerSinkTaskTest.java: ## @@ -1215,17 +1195,13 @@ public void testSuppressCloseErrors() {

Re: [PR] KAFKA-16249; Improve reconciliation state machine [kafka]

2024-03-12 Thread via GitHub
dajac commented on code in PR #15364: URL: https://github.com/apache/kafka/pull/15364#discussion_r1521508636 ## group-coordinator/src/test/java/org/apache/kafka/coordinator/group/GroupMetadataManagerTest.java: ## @@ -931,17 +920,14 @@ public void testGroupEpochBumpWhenNewStaticM

Re: [PR] KAFKA-16249; Improve reconciliation state machine [kafka]

2024-03-12 Thread via GitHub
dajac commented on code in PR #15364: URL: https://github.com/apache/kafka/pull/15364#discussion_r1521507018 ## group-coordinator/src/test/java/org/apache/kafka/coordinator/group/GroupMetadataManagerTest.java: ## @@ -338,7 +338,6 @@ public void testConsumerGroupMemberEpochValida

Re: [PR] KAFKA-14683 Cleanup WorkerSinkTaskTest [kafka]

2024-03-12 Thread via GitHub
OmniaGM commented on code in PR #15506: URL: https://github.com/apache/kafka/pull/15506#discussion_r1521506327 ## connect/runtime/src/test/java/org/apache/kafka/connect/runtime/WorkerSinkTaskTest.java: ## @@ -1215,17 +1195,13 @@ public void testSuppressCloseErrors() {

Re: [PR] MINOR: simplify consumer logic [kafka]

2024-03-12 Thread via GitHub
lianetm commented on code in PR #15519: URL: https://github.com/apache/kafka/pull/15519#discussion_r1521497910 ## clients/src/main/java/org/apache/kafka/clients/consumer/internals/HeartbeatRequestManager.java: ## @@ -529,10 +529,9 @@ public ConsumerGroupHeartbeatRequestData bui

Re: [PR] KAFKA-16249; Improve reconciliation state machine [kafka]

2024-03-12 Thread via GitHub
dajac commented on code in PR #15364: URL: https://github.com/apache/kafka/pull/15364#discussion_r1521495237 ## group-coordinator/src/main/java/org/apache/kafka/coordinator/group/consumer/MemberState.java: ## @@ -0,0 +1,76 @@ +/* + * Licensed to the Apache Software Foundation (A

Re: [PR] KAFKA-16249; Improve reconciliation state machine [kafka]

2024-03-12 Thread via GitHub
dajac commented on code in PR #15364: URL: https://github.com/apache/kafka/pull/15364#discussion_r1521494464 ## group-coordinator/src/main/java/org/apache/kafka/coordinator/group/consumer/CurrentAssignmentBuilder.java: ## @@ -170,72 +127,122 @@ public CurrentAssignmentBuilder wi

Re: [PR] KAFKA-16249; Improve reconciliation state machine [kafka]

2024-03-12 Thread via GitHub
dajac commented on code in PR #15364: URL: https://github.com/apache/kafka/pull/15364#discussion_r1521491629 ## group-coordinator/src/main/java/org/apache/kafka/coordinator/group/consumer/CurrentAssignmentBuilder.java: ## @@ -33,49 +33,6 @@ * The CurrentAssignmentBuilder class

Re: [PR] KAFKA-16249; Improve reconciliation state machine [kafka]

2024-03-12 Thread via GitHub
dajac commented on code in PR #15364: URL: https://github.com/apache/kafka/pull/15364#discussion_r1521490928 ## group-coordinator/src/main/java/org/apache/kafka/coordinator/group/consumer/ConsumerGroupMember.java: ## @@ -528,27 +478,6 @@ public Map> partitionsPendingRevocation(

Re: [PR] KAFKA-16249; Improve reconciliation state machine [kafka]

2024-03-12 Thread via GitHub
dajac commented on code in PR #15364: URL: https://github.com/apache/kafka/pull/15364#discussion_r1521488580 ## group-coordinator/src/main/java/org/apache/kafka/coordinator/group/GroupMetadataManager.java: ## @@ -1796,12 +1834,12 @@ public void onLoaded() {

Re: [PR] KAFKA-16249; Improve reconciliation state machine [kafka]

2024-03-12 Thread via GitHub
dajac commented on code in PR #15364: URL: https://github.com/apache/kafka/pull/15364#discussion_r1521487635 ## group-coordinator/src/main/java/org/apache/kafka/coordinator/group/GroupMetadataManager.java: ## @@ -1400,35 +1440,35 @@ private void cancelConsumerGroupSessionTimeout

Re: [PR] KAFKA-16249; Improve reconciliation state machine [kafka]

2024-03-12 Thread via GitHub
dajac commented on code in PR #15364: URL: https://github.com/apache/kafka/pull/15364#discussion_r1521482723 ## group-coordinator/src/main/java/org/apache/kafka/coordinator/group/GroupMetadataManager.java: ## @@ -1211,13 +1192,99 @@ private CoordinatorResult consumerGr

Re: [PR] KAFKA-16249; Improve reconciliation state machine [kafka]

2024-03-12 Thread via GitHub
dajac commented on code in PR #15364: URL: https://github.com/apache/kafka/pull/15364#discussion_r1521477968 ## group-coordinator/src/main/java/org/apache/kafka/coordinator/group/GroupMetadataManager.java: ## @@ -1211,13 +1192,99 @@ private CoordinatorResult consumerGr

[jira] [Created] (KAFKA-16367) Full ConsumerGroupHeartbeat response must be sent when full request is received

2024-03-12 Thread David Jacot (Jira)
David Jacot created KAFKA-16367: --- Summary: Full ConsumerGroupHeartbeat response must be sent when full request is received Key: KAFKA-16367 URL: https://issues.apache.org/jira/browse/KAFKA-16367 Project

Re: [PR] KAFKA-14683 Cleanup WorkerSinkTaskTest [kafka]

2024-03-12 Thread via GitHub
OmniaGM commented on code in PR #15506: URL: https://github.com/apache/kafka/pull/15506#discussion_r1521477925 ## connect/runtime/src/test/java/org/apache/kafka/connect/runtime/WorkerSinkTaskTest.java: ## @@ -580,12 +570,9 @@ public void testErrorInRebalancePartitionRevocation()

Re: [PR] KAFKA-14683 Cleanup WorkerSinkTaskTest [kafka]

2024-03-12 Thread via GitHub
OmniaGM commented on code in PR #15506: URL: https://github.com/apache/kafka/pull/15506#discussion_r1521475624 ## connect/runtime/src/test/java/org/apache/kafka/connect/runtime/WorkerSinkTaskTest.java: ## @@ -553,12 +547,8 @@ public void testErrorInRebalancePartitionLoss() {

[jira] [Updated] (KAFKA-16293) Test log directory failure in Kraft

2024-03-12 Thread Omnia Ibrahim (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-16293?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Omnia Ibrahim updated KAFKA-16293: -- Fix Version/s: 3.7.1 > Test log directory failure in Kraft > -

[jira] [Updated] (KAFKA-16234) Log directory failure re-creates partitions in another logdir automatically

2024-03-12 Thread Omnia Ibrahim (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-16234?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Omnia Ibrahim updated KAFKA-16234: -- Fix Version/s: 3.7.1 > Log directory failure re-creates partitions in another logdir automatic

[jira] [Updated] (KAFKA-16365) AssignmentsManager mismanages completion notifications

2024-03-12 Thread Omnia Ibrahim (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-16365?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Omnia Ibrahim updated KAFKA-16365: -- Affects Version/s: 3.7.0 > AssignmentsManager mismanages completion notifications > --

[jira] [Updated] (KAFKA-16365) AssignmentsManager mismanages completion notifications

2024-03-12 Thread Omnia Ibrahim (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-16365?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Omnia Ibrahim updated KAFKA-16365: -- Fix Version/s: 3.7.1 > AssignmentsManager mismanages completion notifications > --

[jira] [Updated] (KAFKA-16365) AssignmentsManager mismanages completion notifications

2024-03-12 Thread Omnia Ibrahim (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-16365?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Omnia Ibrahim updated KAFKA-16365: -- Component/s: jbod > AssignmentsManager mismanages completion notifications > -

Re: [PR] KAFKA-16234: Log directory failure re-creates partitions in another logdir automatically [kafka]

2024-03-12 Thread via GitHub
OmniaGM commented on code in PR #15335: URL: https://github.com/apache/kafka/pull/15335#discussion_r1521445120 ## core/src/main/scala/kafka/log/LogManager.scala: ## @@ -355,10 +355,11 @@ class LogManager(logDirs: Seq[File], } else if (logDir.getName.endsWith(UnifiedLog.Stra

Re: [PR] KAFKA-16342: fix getOffsetByMaxTimestamp for compressed records [kafka]

2024-03-12 Thread via GitHub
showuon commented on code in PR #15474: URL: https://github.com/apache/kafka/pull/15474#discussion_r1521398482 ## clients/src/main/java/org/apache/kafka/common/record/MemoryRecordsBuilder.java: ## @@ -243,33 +243,20 @@ public MemoryRecords build() { /** * Get the max

Re: [PR] MINOR: Improve logging in AssignmentsManager [kafka]

2024-03-12 Thread via GitHub
soarez commented on PR #15522: URL: https://github.com/apache/kafka/pull/15522#issuecomment-1991528082 @showuon PTAL -- 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 unsu

Re: [PR] KAFKA-16365: AssignmentsManager callback handling issues [kafka]

2024-03-12 Thread via GitHub
soarez commented on PR #15521: URL: https://github.com/apache/kafka/pull/15521#issuecomment-1991526197 @showuon @OmniaGM @gaurav-narula could you have a look? -- This is an automated message from the Apache Git Service. To respond to the message, please log on to GitHub and use the URL abo

Re: [PR] KAFKA-15206: Fix the flaky RemoteIndexCacheTest.testClose test [kafka]

2024-03-12 Thread via GitHub
nikramakrishnan commented on PR #15523: URL: https://github.com/apache/kafka/pull/15523#issuecomment-1991480460 Thanks for the change, @kamalcph! Just for my knowledge, why is there a possibility that the Shutdownable thread never ran in the CI for this test? -- This is an automated messa

Re: [PR] KAFKA-16341: fix the LogValidator for non-compressed type [kafka]

2024-03-12 Thread via GitHub
johnnychhsu commented on code in PR #15476: URL: https://github.com/apache/kafka/pull/15476#discussion_r1521314724 ## storage/src/main/java/org/apache/kafka/storage/internals/log/LogValidator.java: ## @@ -293,11 +293,11 @@ public ValidationResult assignOffsetsNonCompressed(Long

  1   2   >