[GitHub] [kafka] satishd commented on a change in pull request #11058: KAFKA-12802 Added a file based cache for consumed remote log metadata for each partition to avoid consuming again incase of broke

2021-09-30 Thread GitBox
satishd commented on a change in pull request #11058: URL: https://github.com/apache/kafka/pull/11058#discussion_r719960058 ## File path: storage/src/main/java/org/apache/kafka/server/log/remote/metadata/storage/ConsumerTask.java ## @@ -82,44 +87,148 @@ // User topic part

[GitHub] [kafka] satishd commented on a change in pull request #11058: KAFKA-12802 Added a file based cache for consumed remote log metadata for each partition to avoid consuming again incase of broke

2021-09-30 Thread GitBox
satishd commented on a change in pull request #11058: URL: https://github.com/apache/kafka/pull/11058#discussion_r719951352 ## File path: storage/src/test/java/org/apache/kafka/server/log/remote/metadata/storage/TopicBasedRemoteLogMetadataManagerRestartTest.java ## @@ -0,0 +1,

[GitHub] [kafka] satishd commented on a change in pull request #11058: KAFKA-12802 Added a file based cache for consumed remote log metadata for each partition to avoid consuming again incase of broke

2021-09-30 Thread GitBox
satishd commented on a change in pull request #11058: URL: https://github.com/apache/kafka/pull/11058#discussion_r719950502 ## File path: storage/src/test/java/org/apache/kafka/server/log/remote/metadata/storage/TopicBasedRemoteLogMetadataManagerRestartTest.java ## @@ -0,0 +1,

[GitHub] [kafka] satishd commented on a change in pull request #11058: KAFKA-12802 Added a file based cache for consumed remote log metadata for each partition to avoid consuming again incase of broke

2021-09-30 Thread GitBox
satishd commented on a change in pull request #11058: URL: https://github.com/apache/kafka/pull/11058#discussion_r719940964 ## File path: storage/src/test/java/org/apache/kafka/server/log/remote/metadata/storage/RemoteLogMetadataSnapshotFileTest.java ## @@ -0,0 +1,82 @@ +/* +

[GitHub] [kafka] RivenSun2 commented on pull request #11340: fix issue : KafkaConsumer cannot jump out of the poll method, and the…

2021-09-30 Thread GitBox
RivenSun2 commented on pull request #11340: URL: https://github.com/apache/kafka/pull/11340#issuecomment-931848494 @showuon thanks for your review, i commit new change codes. The test of code changes, I passed the end-to-end test, the consumer is running normally, and the bug has been r

[jira] [Commented] (KAFKA-13336) Migrate StreamsBuilder/Topology class to interfaces and move Topology parameter from KafkaStreams constructor to #start

2021-09-30 Thread Guozhang Wang (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-13336?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17423072#comment-17423072 ] Guozhang Wang commented on KAFKA-13336: --- [~ableegoldman] Thanks for the proposal!

[GitHub] [kafka] RivenSun2 commented on a change in pull request #11340: fix issue : KafkaConsumer cannot jump out of the poll method, and the…

2021-09-30 Thread GitBox
RivenSun2 commented on a change in pull request #11340: URL: https://github.com/apache/kafka/pull/11340#discussion_r719890048 ## File path: clients/src/main/java/org/apache/kafka/clients/consumer/internals/ConsumerCoordinator.java ## @@ -1069,6 +1075,37 @@ private void maybeAu

[GitHub] [kafka] RivenSun2 commented on a change in pull request #11340: fix issue : KafkaConsumer cannot jump out of the poll method, and the…

2021-09-30 Thread GitBox
RivenSun2 commented on a change in pull request #11340: URL: https://github.com/apache/kafka/pull/11340#discussion_r719888357 ## File path: clients/src/main/java/org/apache/kafka/clients/consumer/internals/ConsumerCoordinator.java ## @@ -1016,6 +1018,9 @@ public boolean commit

[GitHub] [kafka] ccding commented on a change in pull request #11345: Allow empty last segment to have missing offset index during recovery

2021-09-30 Thread GitBox
ccding commented on a change in pull request #11345: URL: https://github.com/apache/kafka/pull/11345#discussion_r719866902 ## File path: core/src/main/scala/kafka/log/LogSegment.scala ## @@ -77,8 +77,9 @@ class LogSegment private[log] (val log: FileRecords, timeIndex.resiz

[GitHub] [kafka] kowshik commented on a change in pull request #11345: Allow empty last segment to have missing offset index during recovery

2021-09-30 Thread GitBox
kowshik commented on a change in pull request #11345: URL: https://github.com/apache/kafka/pull/11345#discussion_r719860600 ## File path: core/src/main/scala/kafka/log/LogSegment.scala ## @@ -77,8 +77,9 @@ class LogSegment private[log] (val log: FileRecords, timeIndex.resi

[GitHub] [kafka] junrao commented on a change in pull request #11345: Allow empty last segment to have missing offset index during recovery

2021-09-30 Thread GitBox
junrao commented on a change in pull request #11345: URL: https://github.com/apache/kafka/pull/11345#discussion_r719837715 ## File path: core/src/main/scala/kafka/log/LogSegment.scala ## @@ -77,8 +77,9 @@ class LogSegment private[log] (val log: FileRecords, timeIndex.resiz

[GitHub] [kafka] junrao commented on a change in pull request #11058: KAFKA-12802 Added a file based cache for consumed remote log metadata for each partition to avoid consuming again incase of broker

2021-09-30 Thread GitBox
junrao commented on a change in pull request #11058: URL: https://github.com/apache/kafka/pull/11058#discussion_r719825981 ## File path: storage/src/test/java/org/apache/kafka/server/log/remote/metadata/storage/TopicBasedRemoteLogMetadataManagerRestartTest.java ## @@ -0,0 +1,1

[GitHub] [kafka] gharris1727 commented on a change in pull request #11369: KAFKA-13327, KAFKA-13328, KAFKA-13329: Clean up preflight connector validation

2021-09-30 Thread GitBox
gharris1727 commented on a change in pull request #11369: URL: https://github.com/apache/kafka/pull/11369#discussion_r719774645 ## File path: clients/src/main/java/org/apache/kafka/common/config/ConfigDef.java ## @@ -1116,11 +1119,79 @@ public void ensureValid(String name, Obje

[GitHub] [kafka] kowshik commented on a change in pull request #11345: Allow empty last segment to have missing offset index during recovery

2021-09-30 Thread GitBox
kowshik commented on a change in pull request #11345: URL: https://github.com/apache/kafka/pull/11345#discussion_r719800036 ## File path: core/src/main/scala/kafka/log/LogSegment.scala ## @@ -77,8 +77,9 @@ class LogSegment private[log] (val log: FileRecords, timeIndex.resi

[GitHub] [kafka] C0urante commented on pull request #10907: KAFKA-10000: Exactly-once support for source connectors (KIP-618)

2021-09-30 Thread GitBox
C0urante commented on pull request #10907: URL: https://github.com/apache/kafka/pull/10907#issuecomment-931698206 Resolved merge conflicts. Will try to address review comments soon. -- This is an automated message from the Apache Git Service. To respond to the message, please log on to Gi

[GitHub] [kafka] ccding commented on a change in pull request #11345: Allow empty last segment to have missing offset index during recovery

2021-09-30 Thread GitBox
ccding commented on a change in pull request #11345: URL: https://github.com/apache/kafka/pull/11345#discussion_r719741487 ## File path: core/src/main/scala/kafka/log/LogSegment.scala ## @@ -77,8 +77,9 @@ class LogSegment private[log] (val log: FileRecords, timeIndex.resiz

[GitHub] [kafka] kowshik commented on a change in pull request #11345: Allow empty last segment to have missing offset index during recovery

2021-09-30 Thread GitBox
kowshik commented on a change in pull request #11345: URL: https://github.com/apache/kafka/pull/11345#discussion_r71977 ## File path: core/src/main/scala/kafka/log/LogSegment.scala ## @@ -77,8 +77,9 @@ class LogSegment private[log] (val log: FileRecords, timeIndex.resi

[GitHub] [kafka] kowshik commented on a change in pull request #11345: Allow empty last segment to have missing offset index during recovery

2021-09-30 Thread GitBox
kowshik commented on a change in pull request #11345: URL: https://github.com/apache/kafka/pull/11345#discussion_r71977 ## File path: core/src/main/scala/kafka/log/LogSegment.scala ## @@ -77,8 +77,9 @@ class LogSegment private[log] (val log: FileRecords, timeIndex.resi

[GitHub] [kafka] ijuma merged pull request #11359: MINOR: fix CustomRequestLog ms formatting

2021-09-30 Thread GitBox
ijuma merged pull request #11359: URL: https://github.com/apache/kafka/pull/11359 -- 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...

[GitHub] [kafka] junrao commented on a change in pull request #11345: Allow empty last segment to have missing offset index during recovery

2021-09-30 Thread GitBox
junrao commented on a change in pull request #11345: URL: https://github.com/apache/kafka/pull/11345#discussion_r719715401 ## File path: core/src/main/scala/kafka/log/LogSegment.scala ## @@ -77,8 +77,9 @@ class LogSegment private[log] (val log: FileRecords, timeIndex.resiz

[GitHub] [kafka] ccding commented on a change in pull request #11345: Allow empty last segment to have missing offset index during recovery

2021-09-30 Thread GitBox
ccding commented on a change in pull request #11345: URL: https://github.com/apache/kafka/pull/11345#discussion_r719690581 ## File path: core/src/main/scala/kafka/log/LogSegment.scala ## @@ -77,8 +77,9 @@ class LogSegment private[log] (val log: FileRecords, timeIndex.resiz

[GitHub] [kafka] guozhangwang commented on a change in pull request #11367: MINOR: Do not copy on range for in-memory shared store in stream stream left/out joins

2021-09-30 Thread GitBox
guozhangwang commented on a change in pull request #11367: URL: https://github.com/apache/kafka/pull/11367#discussion_r718905933 ## File path: streams/src/test/java/org/apache/kafka/streams/state/internals/ListValueStoreTest.java ## @@ -136,8 +138,9 @@ public void shouldGetAll

[GitHub] [kafka] junrao commented on a change in pull request #11345: Allow empty last segment to have missing offset index during recovery

2021-09-30 Thread GitBox
junrao commented on a change in pull request #11345: URL: https://github.com/apache/kafka/pull/11345#discussion_r719672498 ## File path: core/src/main/scala/kafka/log/LogSegment.scala ## @@ -77,8 +77,9 @@ class LogSegment private[log] (val log: FileRecords, timeIndex.resiz

[GitHub] [kafka] kowshik commented on a change in pull request #11345: Allow empty last segment to have missing offset index during recovery

2021-09-30 Thread GitBox
kowshik commented on a change in pull request #11345: URL: https://github.com/apache/kafka/pull/11345#discussion_r719669247 ## File path: core/src/main/scala/kafka/log/LogSegment.scala ## @@ -77,8 +77,9 @@ class LogSegment private[log] (val log: FileRecords, timeIndex.resi

[GitHub] [kafka] jolshan commented on a change in pull request #11331: KAFKA-13111: Re-evaluate Fetch Sessions when using topic IDs

2021-09-30 Thread GitBox
jolshan commented on a change in pull request #11331: URL: https://github.com/apache/kafka/pull/11331#discussion_r719668223 ## File path: core/src/main/scala/kafka/server/FetchSession.scala ## @@ -378,53 +378,47 @@ class SessionlessFetchContext(val fetchData: util.Map[TopicPar

[jira] [Commented] (KAFKA-13164) State store is attached to wrong node in the Kafka Streams topology

2021-09-30 Thread Guozhang Wang (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-13164?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17422961#comment-17422961 ] Guozhang Wang commented on KAFKA-13164: --- Adding some thoughts here: 1. With topol

[GitHub] [kafka] jolshan commented on a change in pull request #11331: KAFKA-13111: Re-evaluate Fetch Sessions when using topic IDs

2021-09-30 Thread GitBox
jolshan commented on a change in pull request #11331: URL: https://github.com/apache/kafka/pull/11331#discussion_r719649068 ## File path: core/src/main/scala/kafka/server/KafkaApis.scala ## @@ -870,12 +868,14 @@ class KafkaApis(val requestChannel: RequestChannel, //

[jira] [Updated] (KAFKA-13249) Checkpoints do not contain latest offsets on shutdown when using EOS

2021-09-30 Thread Guozhang Wang (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-13249?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Guozhang Wang updated KAFKA-13249: -- Fix Version/s: 3.0.0 > Checkpoints do not contain latest offsets on shutdown when using EOS >

[jira] [Updated] (KAFKA-13249) Checkpoints do not contain latest offsets on shutdown when using EOS

2021-09-30 Thread Guozhang Wang (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-13249?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Guozhang Wang updated KAFKA-13249: -- Fix Version/s: (was: 3.1.0) > Checkpoints do not contain latest offsets on shutdown when u

[GitHub] [kafka] guozhangwang commented on pull request #11372: [DO NOT MERGE] MINOR: Do not send data on abortable error too

2021-09-30 Thread GitBox
guozhangwang commented on pull request #11372: URL: https://github.com/apache/kafka/pull/11372#issuecomment-931541497 cc @hachikuji @dajac WDYT. -- 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 t

[GitHub] [kafka] guozhangwang commented on pull request #11283: KAFKA-13249: Always update changelog offsets before writing the checkpoint file

2021-09-30 Thread GitBox
guozhangwang commented on pull request #11283: URL: https://github.com/apache/kafka/pull/11283#issuecomment-931541107 SG, will cherry-pick. -- 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 sp

[GitHub] [kafka] guozhangwang commented on pull request #11372: [DO NOT MERGE] MINOR: Do not send data on abortable error too

2021-09-30 Thread GitBox
guozhangwang commented on pull request #11372: URL: https://github.com/apache/kafka/pull/11372#issuecomment-931536992 If this is indeed an issue, then I think the simple fix should be changing the `hasFatalError` to `hasError`. -- This is an automated message from the Apache Git Service.

[GitHub] [kafka] guozhangwang opened a new pull request #11372: [DO NOT MERGE] MINOR: Do not send data on abortable error too

2021-09-30 Thread GitBox
guozhangwang opened a new pull request #11372: URL: https://github.com/apache/kafka/pull/11372 We forbid sending produce request before adding partitions in the txn in the following way: * In `Sender#runOnce`, we would not continue to `sendProducerData` as long as `maybeSendAndPollT

[GitHub] [kafka] vvcephei commented on pull request #11315: KAFKA-10540: Migrate KStream aggregate operations

2021-09-30 Thread GitBox
vvcephei commented on pull request #11315: URL: https://github.com/apache/kafka/pull/11315#issuecomment-931487033 I ran the tests locally and got a pass: `./gradlew clean :streams:testAll` -- This is an automated message from the Apache Git Service. To respond to the message, please log o

[GitHub] [kafka] vvcephei merged pull request #11315: KAFKA-10540: Migrate KStream aggregate operations

2021-09-30 Thread GitBox
vvcephei merged pull request #11315: URL: https://github.com/apache/kafka/pull/11315 -- 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

[GitHub] [kafka] junrao commented on a change in pull request #11345: Allow empty last segment to have missing offset index during recovery

2021-09-30 Thread GitBox
junrao commented on a change in pull request #11345: URL: https://github.com/apache/kafka/pull/11345#discussion_r719567378 ## File path: core/src/main/scala/kafka/log/LogSegment.scala ## @@ -77,8 +77,9 @@ class LogSegment private[log] (val log: FileRecords, timeIndex.resiz

[GitHub] [kafka] heritamas opened a new pull request #11371: MINOR: fix of possible java.nio.file.AccessDeniedException during Con…

2021-09-30 Thread GitBox
heritamas opened a new pull request #11371: URL: https://github.com/apache/kafka/pull/11371 …nect plugin directory scan org.apache.kafka.connect.runtime.isolation.PluginUtils.pluginUrls scans a path and collects plugin candidates from there. However, if a directory is not readable,

[jira] [Created] (KAFKA-13337) Scanning for Connect plugins can fail with AccessDeniedException

2021-09-30 Thread Jira
Tamás Héri created KAFKA-13337: -- Summary: Scanning for Connect plugins can fail with AccessDeniedException Key: KAFKA-13337 URL: https://issues.apache.org/jira/browse/KAFKA-13337 Project: Kafka

[GitHub] [kafka] satishd commented on pull request #11058: KAFKA-12802 Added a file based cache for consumed remote log metadata for each partition to avoid consuming again incase of broker restarts.

2021-09-30 Thread GitBox
satishd commented on pull request #11058: URL: https://github.com/apache/kafka/pull/11058#issuecomment-931422554 Thanks @junrao for the review. Addressed them with the replies and with the latest commit. -- This is an automated message from the Apache Git Service. To respond to the mess

[GitHub] [kafka] satishd commented on a change in pull request #11058: KAFKA-12802 Added a file based cache for consumed remote log metadata for each partition to avoid consuming again incase of broke

2021-09-30 Thread GitBox
satishd commented on a change in pull request #11058: URL: https://github.com/apache/kafka/pull/11058#discussion_r719516930 ## File path: storage/src/main/java/org/apache/kafka/server/log/remote/metadata/storage/RemoteLogSegmentMetadataSnapshot.java ## @@ -0,0 +1,201 @@ +/* +

[GitHub] [kafka] satishd commented on a change in pull request #11058: KAFKA-12802 Added a file based cache for consumed remote log metadata for each partition to avoid consuming again incase of broke

2021-09-30 Thread GitBox
satishd commented on a change in pull request #11058: URL: https://github.com/apache/kafka/pull/11058#discussion_r719295072 ## File path: storage/src/main/java/org/apache/kafka/server/log/remote/metadata/storage/ConsumerTask.java ## @@ -120,6 +171,37 @@ public void run() {

[GitHub] [kafka] satishd commented on a change in pull request #11058: KAFKA-12802 Added a file based cache for consumed remote log metadata for each partition to avoid consuming again incase of broke

2021-09-30 Thread GitBox
satishd commented on a change in pull request #11058: URL: https://github.com/apache/kafka/pull/11058#discussion_r719512519 ## File path: storage/src/main/java/org/apache/kafka/server/log/remote/metadata/storage/ConsumerTask.java ## @@ -240,6 +323,11 @@ public void close() {

[GitHub] [kafka] satishd commented on a change in pull request #11058: KAFKA-12802 Added a file based cache for consumed remote log metadata for each partition to avoid consuming again incase of broke

2021-09-30 Thread GitBox
satishd commented on a change in pull request #11058: URL: https://github.com/apache/kafka/pull/11058#discussion_r719512519 ## File path: storage/src/main/java/org/apache/kafka/server/log/remote/metadata/storage/ConsumerTask.java ## @@ -240,6 +323,11 @@ public void close() {

[GitHub] [kafka] satishd commented on a change in pull request #11058: KAFKA-12802 Added a file based cache for consumed remote log metadata for each partition to avoid consuming again incase of broke

2021-09-30 Thread GitBox
satishd commented on a change in pull request #11058: URL: https://github.com/apache/kafka/pull/11058#discussion_r719511498 ## File path: storage/src/main/java/org/apache/kafka/server/log/remote/metadata/storage/RemoteLogMetadataCache.java ## @@ -161,77 +161,73 @@ public void

[GitHub] [kafka] vvcephei commented on pull request #11315: KAFKA-10540: Migrate KStream aggregate operations

2021-09-30 Thread GitBox
vvcephei commented on pull request #11315: URL: https://github.com/apache/kafka/pull/11315#issuecomment-931374193 Actually, I just looked at the results, and I don't think I'll bother re-triggering the tests. None of those are possibly related: Test Name | Duration | Age -- | -- |

[jira] [Updated] (KAFKA-13334) ERROR Failed to clean up log for __consumer_offsets

2021-09-30 Thread Udaya Kumar (Jira)
[ https://issues.apache.org/jira/browse/KAFKA-13334?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Udaya Kumar updated KAFKA-13334: Priority: Blocker (was: Critical) > ERROR Failed to clean up log for __consumer_offsets > ---

[GitHub] [kafka] satishd commented on a change in pull request #11058: KAFKA-12802 Added a file based cache for consumed remote log metadata for each partition to avoid consuming again incase of broke

2021-09-30 Thread GitBox
satishd commented on a change in pull request #11058: URL: https://github.com/apache/kafka/pull/11058#discussion_r719295072 ## File path: storage/src/main/java/org/apache/kafka/server/log/remote/metadata/storage/ConsumerTask.java ## @@ -120,6 +171,37 @@ public void run() {

[GitHub] [kafka] satishd commented on a change in pull request #11058: KAFKA-12802 Added a file based cache for consumed remote log metadata for each partition to avoid consuming again incase of broke

2021-09-30 Thread GitBox
satishd commented on a change in pull request #11058: URL: https://github.com/apache/kafka/pull/11058#discussion_r719282244 ## File path: storage/src/main/java/org/apache/kafka/server/log/remote/metadata/storage/ConsumerTask.java ## @@ -85,32 +90,78 @@ // Map of remote lo

[GitHub] [kafka] showuon commented on a change in pull request #11211: KAFKA-12960: Enforcing strict retention time for WindowStore and Sess…

2021-09-30 Thread GitBox
showuon commented on a change in pull request #11211: URL: https://github.com/apache/kafka/pull/11211#discussion_r719265099 ## File path: streams/src/test/java/org/apache/kafka/streams/state/internals/RocksDBSessionStoreTest.java ## @@ -60,7 +103,8 @@ public void shouldRemoveE

[GitHub] [kafka] showuon commented on a change in pull request #11211: KAFKA-12960: Enforcing strict retention time for WindowStore and Sess…

2021-09-30 Thread GitBox
showuon commented on a change in pull request #11211: URL: https://github.com/apache/kafka/pull/11211#discussion_r719264146 ## File path: streams/src/main/java/org/apache/kafka/streams/state/internals/WindowedKeyValueIterator.java ## @@ -0,0 +1,57 @@ +/* + * Licensed to the Ap

[GitHub] [kafka] satishd commented on a change in pull request #11058: KAFKA-12802 Added a file based cache for consumed remote log metadata for each partition to avoid consuming again incase of broke

2021-09-30 Thread GitBox
satishd commented on a change in pull request #11058: URL: https://github.com/apache/kafka/pull/11058#discussion_r719226435 ## File path: storage/src/main/java/org/apache/kafka/server/log/remote/metadata/storage/RemoteLogMetadataSnapshotFile.java ## @@ -0,0 +1,239 @@ +/* + * L

[GitHub] [kafka] satishd commented on a change in pull request #11058: KAFKA-12802 Added a file based cache for consumed remote log metadata for each partition to avoid consuming again incase of broke

2021-09-30 Thread GitBox
satishd commented on a change in pull request #11058: URL: https://github.com/apache/kafka/pull/11058#discussion_r719224916 ## File path: storage/src/main/java/org/apache/kafka/server/log/remote/metadata/storage/serialization/RemoteLogMetadataSerde.java ## @@ -39,6 +41,7 @@

[GitHub] [kafka] satishd commented on a change in pull request #11058: KAFKA-12802 Added a file based cache for consumed remote log metadata for each partition to avoid consuming again incase of broke

2021-09-30 Thread GitBox
satishd commented on a change in pull request #11058: URL: https://github.com/apache/kafka/pull/11058#discussion_r719225490 ## File path: storage/src/main/java/org/apache/kafka/server/log/remote/metadata/storage/RemoteLogMetadataSnapshotFile.java ## @@ -0,0 +1,252 @@ +/* + * L

[GitHub] [kafka] satishd commented on a change in pull request #11058: KAFKA-12802 Added a file based cache for consumed remote log metadata for each partition to avoid consuming again incase of broke

2021-09-30 Thread GitBox
satishd commented on a change in pull request #11058: URL: https://github.com/apache/kafka/pull/11058#discussion_r719224916 ## File path: storage/src/main/java/org/apache/kafka/server/log/remote/metadata/storage/serialization/RemoteLogMetadataSerde.java ## @@ -39,6 +41,7 @@

[GitHub] [kafka] showuon commented on pull request #11370: MINOR: remove unneeded size and add lock coarsening to inMemoryKeyValueStore

2021-09-30 Thread GitBox
showuon commented on pull request #11370: URL: https://github.com/apache/kafka/pull/11370#issuecomment-930998456 @mjsax @ableegoldman @guozhangwang , please help review this small PR. Thanks. -- This is an automated message from the Apache Git Service. To respond to the message, please l

[GitHub] [kafka] showuon opened a new pull request #11370: MINOR: remove unneeded size and add lock coarsening to inMemoryKeyValueStore

2021-09-30 Thread GitBox
showuon opened a new pull request #11370: URL: https://github.com/apache/kafka/pull/11370 In https://github.com/apache/kafka/pull/7212, we reverted the change to use `TreeMap` instead of `ConcurrentSkipListMap`, but we forgot to remove the `size` computing codes. Also, I found the `

[GitHub] [kafka] kowshik commented on a change in pull request #11345: Allow empty last segment to have missing offset index during recovery

2021-09-30 Thread GitBox
kowshik commented on a change in pull request #11345: URL: https://github.com/apache/kafka/pull/11345#discussion_r719132048 ## File path: core/src/main/scala/kafka/log/LogSegment.scala ## @@ -77,8 +77,9 @@ class LogSegment private[log] (val log: FileRecords, timeIndex.resi

[GitHub] [kafka] showuon commented on pull request #11234: KAFKA-13212: add support infinite query for session store

2021-09-30 Thread GitBox
showuon commented on pull request #11234: URL: https://github.com/apache/kafka/pull/11234#issuecomment-930892210 @patrickstuedi @vvcephei @guozhangwang , please help review the PR for session store part. Thank you. -- This is an automated message from the Apache Git Service. To respond t

[GitHub] [kafka] showuon commented on a change in pull request #11362: KAFKA-13319: Do not commit empty offsets on producer

2021-09-30 Thread GitBox
showuon commented on a change in pull request #11362: URL: https://github.com/apache/kafka/pull/11362#discussion_r719115383 ## File path: streams/src/test/java/org/apache/kafka/streams/processor/internals/TaskManagerTest.java ## @@ -735,7 +735,6 @@ public void shouldCommitNon