[GitHub] [kafka] RivenSun2 commented on pull request #12010: KAFKA-13793: Add validators for configs that lack validators
RivenSun2 commented on PR #12010: URL: https://github.com/apache/kafka/pull/12010#issuecomment-1092480807 Hi @showuon I'll tease out a KIP later that aims to add it for configs with missing validators. The changes involved in the public interface will be pointed out in the text. Thanks. -- 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.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org
[GitHub] [kafka] RivenSun2 commented on a diff in pull request #12010: KAFKA-13793: Add validators for configs that lack validators
RivenSun2 commented on code in PR #12010: URL: https://github.com/apache/kafka/pull/12010#discussion_r845759271 ## connect/mirror/src/test/java/org/apache/kafka/connect/mirror/MirrorMakerConfigTest.java: ## @@ -49,15 +53,15 @@ public void testClusterConfigProperties() { "clusters", "a, b", "a.bootstrap.servers", "servers-one", "b.bootstrap.servers", "servers-two", -"security.protocol", "SASL", +"security.protocol", "SSL", Review Comment: Since validators are added to `security.protocol`, valid values are: PLAINTEXT, SSL, SASL_PLAINTEXT, SASL_SSL. Make changes here so that testCase can pass normally. -- 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.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org
[GitHub] [kafka] EvansJahja commented on pull request #12012: MINOR: (console consumer) add print.topic property
EvansJahja commented on PR #12012: URL: https://github.com/apache/kafka/pull/12012#issuecomment-1092379826 Hi @dajac, could I ask for your review on this? -- 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.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org
[GitHub] [kafka] showuon commented on a diff in pull request #12010: KAFKA-13793: Add validators for configs that lack validators
showuon commented on code in PR #12010: URL: https://github.com/apache/kafka/pull/12010#discussion_r845685014 ## clients/src/main/java/org/apache/kafka/common/config/ConfigDef.java: ## @@ -1083,6 +1083,26 @@ public String toString() { } } +public static class NonNullAndEmptyString implements Validator { Review Comment: Sorry, `ConfigDef` is public API, needs KIP. ## clients/src/main/java/org/apache/kafka/common/config/SslClientAuth.java: ## @@ -45,4 +56,8 @@ public static SslClientAuth forConfig(String key) { } return null; } + +public static List names() { Review Comment: Sorry, `SslClientAuth` is public API. ## connect/mirror/src/test/java/org/apache/kafka/connect/mirror/MirrorMakerConfigTest.java: ## @@ -49,15 +53,15 @@ public void testClusterConfigProperties() { "clusters", "a, b", "a.bootstrap.servers", "servers-one", "b.bootstrap.servers", "servers-two", -"security.protocol", "SASL", +"security.protocol", "SSL", Review Comment: Why do we need this change? -- 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.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org
[GitHub] [kafka] showuon commented on a diff in pull request #12006: KAFKA-13794: Follow up to fix comparator
showuon commented on code in PR #12006: URL: https://github.com/apache/kafka/pull/12006#discussion_r845677599 ## clients/src/main/java/org/apache/kafka/clients/producer/internals/TransactionManager.java: ## @@ -187,7 +187,7 @@ private void startSequencesAtBeginning(TopicPartition topicPartition, ProducerId private static final Comparator PRODUCER_BATCH_COMPARATOR = (b1, b2) -> { if (b1.baseSequence() < b2.baseSequence()) return -1; else if (b1.baseSequence() > b2.baseSequence()) return 1; -else return b1.equals(b2) ? 0 : 1; +else return b1.equals(b2) ? 0 : Integer.compare(b1.hashCode(), b2.hashCode()); Review Comment: Make sense. Thanks for the update. -- 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.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org
[GitHub] [kafka] ddrid commented on a diff in pull request #12006: KAFKA-13794: Follow up to fix comparator
ddrid commented on code in PR #12006: URL: https://github.com/apache/kafka/pull/12006#discussion_r845672887 ## clients/src/main/java/org/apache/kafka/clients/producer/internals/TransactionManager.java: ## @@ -187,7 +187,7 @@ private void startSequencesAtBeginning(TopicPartition topicPartition, ProducerId private static final Comparator PRODUCER_BATCH_COMPARATOR = (b1, b2) -> { if (b1.baseSequence() < b2.baseSequence()) return -1; else if (b1.baseSequence() > b2.baseSequence()) return 1; -else return b1.equals(b2) ? 0 : 1; +else return b1.equals(b2) ? 0 : Integer.compare(b1.hashCode(), b2.hashCode()); Review Comment: @artemlivshits done, thanks for your suggestion -- 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.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org
[GitHub] [kafka] dengziming commented on a diff in pull request #11681: KAFKA-8785: fix request timeout by waiting for metadata cache up-to-date
dengziming commented on code in PR #11681: URL: https://github.com/apache/kafka/pull/11681#discussion_r845640296 ## core/src/test/java/kafka/testkit/KafkaClusterTestKit.java: ## @@ -373,13 +373,19 @@ public void startup() throws ExecutionException, InterruptedException { /** * Wait for a controller to mark all the brokers as ready (registered and unfenced). + * And also wait for the metadata cache up-to-date in each broker server. */ public void waitForReadyBrokers() throws ExecutionException, InterruptedException { // We can choose any controller, not just the active controller. // If we choose a standby controller, we will wait slightly longer. ControllerServer controllerServer = controllers.values().iterator().next(); Review Comment: It seems we no longer have to call `controller.waitForReadyBrokers()` if we will wait for metadataCache to catch up. -- 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.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org
[jira] [Created] (KAFKA-13807) Ensure that we can set log.flush.interval.ms with IncrementalAlterConfigs
Colin McCabe created KAFKA-13807: Summary: Ensure that we can set log.flush.interval.ms with IncrementalAlterConfigs Key: KAFKA-13807 URL: https://issues.apache.org/jira/browse/KAFKA-13807 Project: Kafka Issue Type: Bug Reporter: Colin McCabe -- This message was sent by Atlassian Jira (v8.20.1#820001)
[GitHub] [kafka] tmbull commented on pull request #8103: KAFKA-7061: KIP-280 Enhanced log compaction
tmbull commented on PR #8103: URL: https://github.com/apache/kafka/pull/8103#issuecomment-1092308800 Is there any update on this? -- 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.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org
[GitHub] [kafka] mjsax commented on a diff in pull request #11896: KAFKA-13785: [Emit final][5/N] emit final for TimeWindowedKStreamImpl
mjsax commented on code in PR #11896: URL: https://github.com/apache/kafka/pull/11896#discussion_r845465254 ## streams/src/main/java/org/apache/kafka/streams/kstream/internals/KStreamWindowAggregate.java: ## @@ -80,22 +109,54 @@ public void enableSendingOldValues() { private TimestampedWindowStore windowStore; private TimestampedTupleForwarder, VAgg> tupleForwarder; private Sensor droppedRecordsSensor; +private Sensor emittedRecordsSensor; +private Sensor emitFinalLatencySensor; private long observedStreamTime = ConsumerRecord.NO_TIMESTAMP; +private long lastEmitCloseTime = ConsumerRecord.NO_TIMESTAMP; +private InternalProcessorContext, Change> internalProcessorContext; +private final TimeTracker timeTracker = new TimeTracker(); +private final Time time = Time.SYSTEM; @Override public void init(final ProcessorContext, Change> context) { super.init(context); -final InternalProcessorContext, Change> internalProcessorContext = -(InternalProcessorContext, Change>) context; +internalProcessorContext = (InternalProcessorContext, Change>) context; final StreamsMetricsImpl metrics = internalProcessorContext.metrics(); final String threadId = Thread.currentThread().getName(); droppedRecordsSensor = droppedRecordsSensor(threadId, context.taskId().toString(), metrics); +emittedRecordsSensor = emittedRecordsSensor(threadId, context.taskId().toString(), metrics); +emitFinalLatencySensor = emitFinalLatencySensor(threadId, context.taskId().toString(), metrics); windowStore = context.getStateStore(storeName); -tupleForwarder = new TimestampedTupleForwarder<>( -windowStore, -context, -new TimestampedCacheFlushListener<>(context), -sendOldValues); + +if (emitStrategy.type() == StrategyType.ON_WINDOW_CLOSE) { +// Don't set flush lister which emit cache results +tupleForwarder = new TimestampedTupleForwarder<>( +windowStore, +context, +sendOldValues); +} else { +tupleForwarder = new TimestampedTupleForwarder<>( +windowStore, +context, +new TimestampedCacheFlushListener<>(context), +sendOldValues); +} + +log.info("EmitStrategy=" + emitStrategy.type()); Review Comment: Might debug level be better? ## streams/src/main/java/org/apache/kafka/streams/kstream/EmitStrategy.java: ## @@ -0,0 +1,65 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one or more + * contributor license agreements. See the NOTICE file distributed with + * this work for additional information regarding copyright ownership. + * The ASF licenses this file to You under the Apache License, Version 2.0 + * (the "License"); you may not use this file except in compliance with + * the License. You may obtain a copy of the License at + * + *http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ +package org.apache.kafka.streams.kstream; + +import org.apache.kafka.streams.kstream.internals.UnlimitedWindow; +import org.apache.kafka.streams.kstream.internals.emitstrategy.WindowCloseStrategy; +import org.apache.kafka.streams.kstream.internals.emitstrategy.WindowUpdateStrategy; + +/** + * This interface controls the strategy that can be used to control how we emit results in a processor. + */ +public interface EmitStrategy { + +enum StrategyType { +ON_WINDOW_CLOSE, +ON_WINDOW_UPDATE +} + +/** + * Returns the strategy type + * @return Emit strategy type + */ +StrategyType type(); + +/** + * This strategy indicates that the aggregated result for a window will only be outputted when the + * window closes instead of when there's an update to the window. + * + * This strategy should only be used for window which can close. For example, it doesn't make sense + * to be used with {@link UnlimitedWindow} Review Comment: What happens if it's used anyway? (nit: missing `.` at the end of the sentence) ## streams/src/main/java/org/apache/kafka/streams/kstream/internals/KStreamWindowAggregate.java: ## @@ -80,22 +109,54 @@ public void enableSendingOldValues() { private TimestampedWindowStore windowStore; private TimestampedTupleForwarder, VAgg> t
[jira] [Commented] (KAFKA-7509) Kafka Connect logs unnecessary warnings about unused configurations
[ https://issues.apache.org/jira/browse/KAFKA-7509?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17519199#comment-17519199 ] Matthias J. Sax commented on KAFKA-7509: Thanks for staying on top of this. The filtering code is a partial solution to the problem. Also the prefixes like `producer.` et al: yes they help. Not sure if they solve all issues. KAFKA-6793 actually illustrates the general problem we tried to solve. However, IIRC KAFKA-6793 in particular was triggered by `StreamsConfig#retries` that we needed to pass though the admin client into `StreamsPartitionAssignor`. `StreamConfig#retries` was deprecated (it's still in the code, but unused and not set on the `AdiminConfig` any longer) via KIP-572 [https://cwiki.apache.org/confluence/display/KAFKA/KIP-572%3A+Improve+timeouts+and+retries+in+Kafka+Streams|https://cwiki.apache.org/confluence/display/KAFKA/KIP-572%3A+Improve+timeouts+and+retries+in+Kafka+Streams(] -- thus, KAFKA-6793 might effectively resolved as a side effect, but only because we stopped passing in an unrecognized config – thus, the underlying principle issue is still there. > Kafka Connect logs unnecessary warnings about unused configurations > --- > > Key: KAFKA-7509 > URL: https://issues.apache.org/jira/browse/KAFKA-7509 > Project: Kafka > Issue Type: Improvement > Components: clients, KafkaConnect >Affects Versions: 0.10.2.0 >Reporter: Randall Hauch >Assignee: Chris Egerton >Priority: Major > > When running Connect, the logs contain quite a few warnings about "The > configuration '{}' was supplied but isn't a known config." This occurs when > Connect creates producers, consumers, and admin clients, because the > AbstractConfig is logging unused configuration properties upon construction. > It's complicated by the fact that the Producer, Consumer, and AdminClient all > create their own AbstractConfig instances within the constructor, so we can't > even call its {{ignore(String key)}} method. > See also KAFKA-6793 for a similar issue with Streams. > There are no arguments in the Producer, Consumer, or AdminClient constructors > to control whether the configs log these warnings, so a simpler workaround > is to only pass those configuration properties to the Producer, Consumer, and > AdminClient that the ProducerConfig, ConsumerConfig, and AdminClientConfig > configdefs know about. -- This message was sent by Atlassian Jira (v8.20.1#820001)
[jira] [Commented] (KAFKA-9296) Correlation id for response () does not match request ()
[ https://issues.apache.org/jira/browse/KAFKA-9296?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17519194#comment-17519194 ] Jun Rao commented on KAFKA-9296: [~vongosling] : It would be useful to provide a bit more detail on when the issue occurred. For example, did it happen on plain or SSL socket? Were there any disconnects either on the server side or the client? > Correlation id for response () does not match request () > > > Key: KAFKA-9296 > URL: https://issues.apache.org/jira/browse/KAFKA-9296 > Project: Kafka > Issue Type: Bug > Components: clients, producer >Affects Versions: 0.11.0.2 > Environment: Flink on k8s >Reporter: Enhon Bryant >Priority: Blocker > Labels: kafka, producer > > The Kafka client and broker I use are both version 0.11.0.2. I use Kafka's > producer to write data to broker. I encountered the following exceptions. > 2019-12-12 18:12:46,821 ERROR > org.apache.kafka.clients.producer.internals.Sender - Uncaught error in kafka > producer I/O thread: > java.lang.IllegalStateException: Correlation id for response (11715816) does > not match request (11715804), request header: > \{api_key=0,api_version=3,correlation_id=11715804,client_id=producer-3} > at org.apache.kafka.clients.NetworkClient.correlate(NetworkClient.java:752) > at > org.apache.kafka.clients.NetworkClient.parseStructMaybeUpdateThrottleTimeMetrics(NetworkClient.java:561) > at > org.apache.kafka.clients.NetworkClient.handleCompletedReceives(NetworkClient.java:657) > at org.apache.kafka.clients.NetworkClient.poll(NetworkClient.java:442) > at org.apache.kafka.clients.producer.internals.Sender.run(Sender.java:224) > at org.apache.kafka.clients.producer.internals.Sender.run(Sender.java:162) > at java.lang.Thread.run(Thread.java:748) -- This message was sent by Atlassian Jira (v8.20.1#820001)
[GitHub] [kafka] ijuma opened a new pull request, #12014: MINOR: Fix support for custom commit ids in the build
ijuma opened a new pull request, #12014: URL: https://github.com/apache/kafka/pull/12014 This regressed in ca375d8004c1 due to a typo. We need tests for our builds. :) ### Committer Checklist (excluded from commit message) - [ ] Verify design and implementation - [ ] Verify test coverage and CI build status - [ ] Verify documentation (including upgrade notes) -- 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.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org
[jira] [Commented] (KAFKA-12909) Allow users to opt-into spurious left/outer stream-stream join improvement
[ https://issues.apache.org/jira/browse/KAFKA-12909?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17519175#comment-17519175 ] Matthias J. Sax commented on KAFKA-12909: - Not that _inner_ join result are still instant, even if you use `leftJoin` or `outerJoin` – there are no plan to emit left/right join results eagerly again, but make the new methods mandatory. Why would you want left/right join result to be emitted eagerly (and thus, potentially incorrectly?) > Allow users to opt-into spurious left/outer stream-stream join improvement > -- > > Key: KAFKA-12909 > URL: https://issues.apache.org/jira/browse/KAFKA-12909 > Project: Kafka > Issue Type: Improvement > Components: streams >Reporter: Matthias J. Sax >Assignee: Matthias J. Sax >Priority: Blocker > Fix For: 3.1.0 > > > https://issues.apache.org/jira/browse/KAFKA-10847 improves left/outer > stream-stream join, by not emitting left/outer results eagerly, but only > after the grace period passed. > While this change is desired, there is an issue with regard to upgrades: if > users don't specify a grace period, we fall back to a 24h default. Thus, > left/outer join results would only be emitted 24h after the join window end. > This change in behavior could break existing applications when upgrading to > 3.0.0 release. – And even if users do set a grace period explicitly, it's > still unclear if the new delayed output behavior would work for them. > Thus, we propose to disable the fix of KAFAK-10847 by default, and let user > opt-into the fix explicitly instead. > To allow users to enable the fix, we want to piggy-back on KIP-633 > (https://issues.apache.org/jira/browse/KAFKA-8613) that deprecated the > existing `JoinWindows.of()` and `JoinWindows#grace()` methods in favor of > `JoinWindows.ofSizeAndGrace()` – if users don't update their code, we would > keep the fix disabled, and thus, if users upgrade their app nothing changes. > Only if users switch to the new `ofSizeAndGrace()` API, we enable the fix and > thus give users the opportunity to opt-in expliclity and pick an appropriate > grace period for their application. -- This message was sent by Atlassian Jira (v8.20.1#820001)
[GitHub] [kafka] rittikaadhikari commented on a diff in pull request #12005: KAFKA-13803: Refactor Leader API Access
rittikaadhikari commented on code in PR #12005: URL: https://github.com/apache/kafka/pull/12005#discussion_r845546314 ## core/src/main/scala/kafka/server/LeaderEndPoint.scala: ## @@ -0,0 +1,66 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one or more + * contributor license agreements. See the NOTICE file distributed with + * this work for additional information regarding copyright ownership. + * The ASF licenses this file to You under the Apache License, Version 2.0 + * (the "License"); you may not use this file except in compliance with + * the License. You may obtain a copy of the License at + * + *http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ + +package kafka.server + +import kafka.api.{KAFKA_0_10_1_IV2, KAFKA_0_11_0_IV0, KAFKA_2_0_IV0, KAFKA_2_0_IV1, KAFKA_2_1_IV1, KAFKA_2_2_IV1, KAFKA_2_3_IV1, KAFKA_2_8_IV0, KAFKA_3_0_IV1} +import kafka.utils.Logging +import org.apache.kafka.common.TopicPartition +import org.apache.kafka.common.requests.FetchRequest +import org.apache.kafka.common.message.OffsetForLeaderEpochResponseData.EpochEndOffset +import org.apache.kafka.common.message.{FetchResponseData, OffsetForLeaderEpochRequestData} + +import scala.collection.Map + +trait LeaderEndPoint extends Logging { + val brokerConfig: KafkaConfig + val endpoint: BlockingSend = null Review Comment: Do you think it would make more sense to make the trait an abstract class, so that these additional parameters could be added to the ctor? -- 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.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org
[jira] [Updated] (KAFKA-13803) Refactor Leader API Access
[ https://issues.apache.org/jira/browse/KAFKA-13803?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Rittika Adhikari updated KAFKA-13803: - Description: Currently, AbstractFetcherThread has a series of protected APIs which control access to the Leader. ReplicaFetcherThread and ReplicaAlterLogDirsThread respectively override these protected APIs and handle access to the Leader in a remote broker leader and a local leader context. We propose to move these protected APIs to a LeaderEndPoint interface, which will serve all fetches from the Leader. We will implement a RemoteLeaderEndPoint and a LocalLeaderEndPoint accordingly. This change will greatly simplify our existing follower fetch code. was: Currently, AbstractFetcherThread has a series of protected APIs which control access to the Leader. ReplicaFetcherThread and ReplicaAlterLogDirsThread respectively override these protected APIs and handle access to the Leader in a remote and local object store context. We propose to move these protected APIs to a LeaderEndPoint interface, which will serve all fetches from the Leader. We will implement a RemoteLeaderEndPoint and a LocalLeaderEndPoint accordingly. This change will greatly simplify our existing follower fetch code. > Refactor Leader API Access > -- > > Key: KAFKA-13803 > URL: https://issues.apache.org/jira/browse/KAFKA-13803 > Project: Kafka > Issue Type: Improvement >Reporter: Rittika Adhikari >Assignee: Rittika Adhikari >Priority: Major > > Currently, AbstractFetcherThread has a series of protected APIs which control > access to the Leader. ReplicaFetcherThread and ReplicaAlterLogDirsThread > respectively override these protected APIs and handle access to the Leader in > a remote broker leader and a local leader context. > We propose to move these protected APIs to a LeaderEndPoint interface, which > will serve all fetches from the Leader. We will implement a > RemoteLeaderEndPoint and a LocalLeaderEndPoint accordingly. This change will > greatly simplify our existing follower fetch code. -- This message was sent by Atlassian Jira (v8.20.1#820001)
[GitHub] [kafka] rittikaadhikari commented on a diff in pull request #12005: KAFKA-13803: Refactor Leader API Access
rittikaadhikari commented on code in PR #12005: URL: https://github.com/apache/kafka/pull/12005#discussion_r845546314 ## core/src/main/scala/kafka/server/LeaderEndPoint.scala: ## @@ -0,0 +1,66 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one or more + * contributor license agreements. See the NOTICE file distributed with + * this work for additional information regarding copyright ownership. + * The ASF licenses this file to You under the Apache License, Version 2.0 + * (the "License"); you may not use this file except in compliance with + * the License. You may obtain a copy of the License at + * + *http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ + +package kafka.server + +import kafka.api.{KAFKA_0_10_1_IV2, KAFKA_0_11_0_IV0, KAFKA_2_0_IV0, KAFKA_2_0_IV1, KAFKA_2_1_IV1, KAFKA_2_2_IV1, KAFKA_2_3_IV1, KAFKA_2_8_IV0, KAFKA_3_0_IV1} +import kafka.utils.Logging +import org.apache.kafka.common.TopicPartition +import org.apache.kafka.common.requests.FetchRequest +import org.apache.kafka.common.message.OffsetForLeaderEpochResponseData.EpochEndOffset +import org.apache.kafka.common.message.{FetchResponseData, OffsetForLeaderEpochRequestData} + +import scala.collection.Map + +trait LeaderEndPoint extends Logging { + val brokerConfig: KafkaConfig + val endpoint: BlockingSend = null Review Comment: Do you think it would make sense to pass in the `endpoint` as an opt for the fetch functions, by default set to `Option.empty`? -- 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.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org
[jira] [Commented] (KAFKA-7491) Kafka streams and Kafka streams test utils have split packages
[ https://issues.apache.org/jira/browse/KAFKA-7491?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17519124#comment-17519124 ] Karsten Schnitter commented on KAFKA-7491: -- Hi, this issue has not seen much action in quite some time. I am experiencing the same issue. I tried patch-module in the pom.xml but to no avail: {code:xml} org.apache.maven.plugins maven-compiler-plugin test-compile process-test-sources testCompile --patch-module kafka.streams=${settings.localRepository}/org/apache/kafka/kafka-streams-test-utils/3.0.1/kafka-streams-test-utils-3.0.1.jar {code} That way, I got around the "missing symbol" issue, that is caused by the test-util sitting in the same package on the classpath, where it is abandoned due to the automatic kafka-streams module. But compilation still errors with an incompatible type error for a TestRecord. The offending code in my project is: {code:java} private TestRecord readOutput() { return outputTopic.readRecord(); } {code} The error message is: {noformat} incompatible types: org.apache.kafka.streams.test.TestRecord cannot be converted to org.apache.kafka.streams.test.TestRecord {noformat} I cannot say for sure, if my own {{QuotaStatisticsMessage}} may cause the problem, but I doubt it, since there are no failures at other places. I think, the {{TestRecord}} might be incompatible, maybe due to the {{--patch-module}}. Is there any idea, how to proceed. One way would be to release a second test-util with different package names. > Kafka streams and Kafka streams test utils have split packages > -- > > Key: KAFKA-7491 > URL: https://issues.apache.org/jira/browse/KAFKA-7491 > Project: Kafka > Issue Type: Bug > Components: streams >Reporter: Robin Van Praet >Priority: Major > > When trying to test a Kafka Streams application using JDK 9+, using the > module path, (test) compilation errors occur. > The TopologyTestDriver cannot be located in kafka-streams-test-utils because > package 'org.apache.kafka.streams' is already provided by module > kafka-streams. > Please make sure that packages are not re-used between production libraries > and test util libraries. -- This message was sent by Atlassian Jira (v8.20.1#820001)
[GitHub] [kafka] dajac commented on a diff in pull request #12005: KAFKA-13803: Refactor Leader API Access
dajac commented on code in PR #12005: URL: https://github.com/apache/kafka/pull/12005#discussion_r845478395 ## core/src/main/scala/kafka/server/LeaderEndPoint.scala: ## @@ -0,0 +1,66 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one or more + * contributor license agreements. See the NOTICE file distributed with + * this work for additional information regarding copyright ownership. + * The ASF licenses this file to You under the Apache License, Version 2.0 + * (the "License"); you may not use this file except in compliance with + * the License. You may obtain a copy of the License at + * + *http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ + +package kafka.server + +import kafka.api.{KAFKA_0_10_1_IV2, KAFKA_0_11_0_IV0, KAFKA_2_0_IV0, KAFKA_2_0_IV1, KAFKA_2_1_IV1, KAFKA_2_2_IV1, KAFKA_2_3_IV1, KAFKA_2_8_IV0, KAFKA_3_0_IV1} +import kafka.utils.Logging +import org.apache.kafka.common.TopicPartition +import org.apache.kafka.common.requests.FetchRequest +import org.apache.kafka.common.message.OffsetForLeaderEpochResponseData.EpochEndOffset +import org.apache.kafka.common.message.{FetchResponseData, OffsetForLeaderEpochRequestData} + +import scala.collection.Map + +trait LeaderEndPoint extends Logging { + val brokerConfig: KafkaConfig Review Comment: Passing it in the constructor of `RemoteLeaderEndPoint` is fine. I just think that providing it via the interface to other components is weird. -- 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.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org
[GitHub] [kafka] dajac commented on a diff in pull request #12005: KAFKA-13803: Refactor Leader API Access
dajac commented on code in PR #12005: URL: https://github.com/apache/kafka/pull/12005#discussion_r845476205 ## core/src/main/scala/kafka/server/ReplicaFetcherThread.scala: ## @@ -213,62 +174,6 @@ class ReplicaFetcherThread(name: String, "equal or larger than your settings for max.message.bytes, both at a broker and topic level.") } - - override protected def fetchFromLeader(fetchRequest: FetchRequest.Builder): Map[TopicPartition, FetchData] = { -val clientResponse = try { - leaderEndpoint.sendRequest(fetchRequest) -} catch { - case t: Throwable => -fetchSessionHandler.handleError(t) -throw t -} -val fetchResponse = clientResponse.responseBody.asInstanceOf[FetchResponse] -if (!fetchSessionHandler.handleResponse(fetchResponse, clientResponse.requestHeader().apiVersion())) { - // If we had a session topic ID related error, throw it, otherwise return an empty fetch data map. - if (fetchResponse.error == Errors.FETCH_SESSION_TOPIC_ID_ERROR) { -throw Errors.forCode(fetchResponse.error().code()).exception() - } else { -Map.empty - } -} else { - fetchResponse.responseData(fetchSessionHandler.sessionTopicNames, clientResponse.requestHeader().apiVersion()).asScala -} - } - - override protected def fetchEarliestOffsetFromLeader(topicPartition: TopicPartition, currentLeaderEpoch: Int): Long = { -fetchOffsetFromLeader(topicPartition, currentLeaderEpoch, ListOffsetsRequest.EARLIEST_TIMESTAMP) - } - - override protected def fetchLatestOffsetFromLeader(topicPartition: TopicPartition, currentLeaderEpoch: Int): Long = { -fetchOffsetFromLeader(topicPartition, currentLeaderEpoch, ListOffsetsRequest.LATEST_TIMESTAMP) - } - - private def fetchOffsetFromLeader(topicPartition: TopicPartition, currentLeaderEpoch: Int, earliestOrLatest: Long): Long = { -val topic = new ListOffsetsTopic() - .setName(topicPartition.topic) - .setPartitions(Collections.singletonList( - new ListOffsetsPartition() -.setPartitionIndex(topicPartition.partition) -.setCurrentLeaderEpoch(currentLeaderEpoch) -.setTimestamp(earliestOrLatest))) -val requestBuilder = ListOffsetsRequest.Builder.forReplica(listOffsetRequestVersion, replicaId) - .setTargetTimes(Collections.singletonList(topic)) - -val clientResponse = leaderEndpoint.sendRequest(requestBuilder) -val response = clientResponse.responseBody.asInstanceOf[ListOffsetsResponse] -val responsePartition = response.topics.asScala.find(_.name == topicPartition.topic).get - .partitions.asScala.find(_.partitionIndex == topicPartition.partition).get - - Errors.forCode(responsePartition.errorCode) match { - case Errors.NONE => -if (brokerConfig.interBrokerProtocolVersion >= KAFKA_0_10_1_IV2) - responsePartition.offset -else - responsePartition.oldStyleOffsets.get(0) - case error => throw error.exception -} - } - override def buildFetch(partitionMap: Map[TopicPartition, PartitionFetchState]): ResultWithPartitions[Option[ReplicaFetch]] = { Review Comment: Have we considered pushing this one to the trait as well? It is implementation is very dependant on the type of the endpoint so that might be interesting. This is also the only place where `fetchSessionHandler` is used in this class. We could have it all handled in the endpoint. -- 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.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org
[GitHub] [kafka] rittikaadhikari commented on a diff in pull request #12005: KAFKA-13803: Refactor Leader API Access
rittikaadhikari commented on code in PR #12005: URL: https://github.com/apache/kafka/pull/12005#discussion_r845476760 ## core/src/main/scala/kafka/server/LeaderEndPoint.scala: ## @@ -0,0 +1,66 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one or more + * contributor license agreements. See the NOTICE file distributed with + * this work for additional information regarding copyright ownership. + * The ASF licenses this file to You under the Apache License, Version 2.0 + * (the "License"); you may not use this file except in compliance with + * the License. You may obtain a copy of the License at + * + *http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ + +package kafka.server + +import kafka.api.{KAFKA_0_10_1_IV2, KAFKA_0_11_0_IV0, KAFKA_2_0_IV0, KAFKA_2_0_IV1, KAFKA_2_1_IV1, KAFKA_2_2_IV1, KAFKA_2_3_IV1, KAFKA_2_8_IV0, KAFKA_3_0_IV1} +import kafka.utils.Logging +import org.apache.kafka.common.TopicPartition +import org.apache.kafka.common.requests.FetchRequest +import org.apache.kafka.common.message.OffsetForLeaderEpochResponseData.EpochEndOffset +import org.apache.kafka.common.message.{FetchResponseData, OffsetForLeaderEpochRequestData} + +import scala.collection.Map + +trait LeaderEndPoint extends Logging { + val brokerConfig: KafkaConfig Review Comment: Yeah, I think only `RemoteLeaderEndPoint` uses the `brokerConfig` and the `*RequestVersion` functions. It might make more sense to optionally pass in the `brokerConfig` and the `*RequestVersion` to the functions that use it in `RemoteLeaderEndPoint` (i.e., all of the `fetch*Offsets` functions) -- 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.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org
[GitHub] [kafka] dajac commented on a diff in pull request #12005: KAFKA-13803: Refactor Leader API Access
dajac commented on code in PR #12005: URL: https://github.com/apache/kafka/pull/12005#discussion_r845476205 ## core/src/main/scala/kafka/server/ReplicaFetcherThread.scala: ## @@ -213,62 +174,6 @@ class ReplicaFetcherThread(name: String, "equal or larger than your settings for max.message.bytes, both at a broker and topic level.") } - - override protected def fetchFromLeader(fetchRequest: FetchRequest.Builder): Map[TopicPartition, FetchData] = { -val clientResponse = try { - leaderEndpoint.sendRequest(fetchRequest) -} catch { - case t: Throwable => -fetchSessionHandler.handleError(t) -throw t -} -val fetchResponse = clientResponse.responseBody.asInstanceOf[FetchResponse] -if (!fetchSessionHandler.handleResponse(fetchResponse, clientResponse.requestHeader().apiVersion())) { - // If we had a session topic ID related error, throw it, otherwise return an empty fetch data map. - if (fetchResponse.error == Errors.FETCH_SESSION_TOPIC_ID_ERROR) { -throw Errors.forCode(fetchResponse.error().code()).exception() - } else { -Map.empty - } -} else { - fetchResponse.responseData(fetchSessionHandler.sessionTopicNames, clientResponse.requestHeader().apiVersion()).asScala -} - } - - override protected def fetchEarliestOffsetFromLeader(topicPartition: TopicPartition, currentLeaderEpoch: Int): Long = { -fetchOffsetFromLeader(topicPartition, currentLeaderEpoch, ListOffsetsRequest.EARLIEST_TIMESTAMP) - } - - override protected def fetchLatestOffsetFromLeader(topicPartition: TopicPartition, currentLeaderEpoch: Int): Long = { -fetchOffsetFromLeader(topicPartition, currentLeaderEpoch, ListOffsetsRequest.LATEST_TIMESTAMP) - } - - private def fetchOffsetFromLeader(topicPartition: TopicPartition, currentLeaderEpoch: Int, earliestOrLatest: Long): Long = { -val topic = new ListOffsetsTopic() - .setName(topicPartition.topic) - .setPartitions(Collections.singletonList( - new ListOffsetsPartition() -.setPartitionIndex(topicPartition.partition) -.setCurrentLeaderEpoch(currentLeaderEpoch) -.setTimestamp(earliestOrLatest))) -val requestBuilder = ListOffsetsRequest.Builder.forReplica(listOffsetRequestVersion, replicaId) - .setTargetTimes(Collections.singletonList(topic)) - -val clientResponse = leaderEndpoint.sendRequest(requestBuilder) -val response = clientResponse.responseBody.asInstanceOf[ListOffsetsResponse] -val responsePartition = response.topics.asScala.find(_.name == topicPartition.topic).get - .partitions.asScala.find(_.partitionIndex == topicPartition.partition).get - - Errors.forCode(responsePartition.errorCode) match { - case Errors.NONE => -if (brokerConfig.interBrokerProtocolVersion >= KAFKA_0_10_1_IV2) - responsePartition.offset -else - responsePartition.oldStyleOffsets.get(0) - case error => throw error.exception -} - } - override def buildFetch(partitionMap: Map[TopicPartition, PartitionFetchState]): ResultWithPartitions[Option[ReplicaFetch]] = { Review Comment: Have we considered pushing this one to the trait as well? It is implementation is very dependant on the type of the endpoint so that might be interesting. -- 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.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org
[GitHub] [kafka] jeqo commented on pull request #11999: [MINOR] fix(streams): align variable names
jeqo commented on PR #11999: URL: https://github.com/apache/kafka/pull/11999#issuecomment-1092096823 cc @vvcephei -- 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.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org
[GitHub] [kafka] dajac commented on a diff in pull request #12005: KAFKA-13803: Refactor Leader API Access
dajac commented on code in PR #12005: URL: https://github.com/apache/kafka/pull/12005#discussion_r845468553 ## core/src/main/scala/kafka/server/LeaderEndPoint.scala: ## @@ -0,0 +1,66 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one or more + * contributor license agreements. See the NOTICE file distributed with + * this work for additional information regarding copyright ownership. + * The ASF licenses this file to You under the Apache License, Version 2.0 + * (the "License"); you may not use this file except in compliance with + * the License. You may obtain a copy of the License at + * + *http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ + +package kafka.server + +import kafka.api.{KAFKA_0_10_1_IV2, KAFKA_0_11_0_IV0, KAFKA_2_0_IV0, KAFKA_2_0_IV1, KAFKA_2_1_IV1, KAFKA_2_2_IV1, KAFKA_2_3_IV1, KAFKA_2_8_IV0, KAFKA_3_0_IV1} +import kafka.utils.Logging +import org.apache.kafka.common.TopicPartition +import org.apache.kafka.common.requests.FetchRequest +import org.apache.kafka.common.message.OffsetForLeaderEpochResponseData.EpochEndOffset +import org.apache.kafka.common.message.{FetchResponseData, OffsetForLeaderEpochRequestData} + +import scala.collection.Map + +trait LeaderEndPoint extends Logging { + val brokerConfig: KafkaConfig Review Comment: I also wonder about the config. The *Threads have also a reference to the KafkaConfig so I don't really see the value of providing it via the interface as well. What was the reasoning? -- 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.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org
[GitHub] [kafka] cadonna commented on pull request #11681: KAFKA-8785: fix request timeout by waiting for metadata cache up-to-date
cadonna commented on PR #11681: URL: https://github.com/apache/kafka/pull/11681#issuecomment-1092089272 @showuon @dengziming @hachikuji @dajac What is the status of this PR? This PR would resolve a blocker for the 3.2.0 release. Would be great if we could merge this PR as soon as possible? -- 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.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org
[GitHub] [kafka] dajac commented on a diff in pull request #12005: KAFKA-13803: Refactor Leader API Access
dajac commented on code in PR #12005: URL: https://github.com/apache/kafka/pull/12005#discussion_r845466834 ## core/src/main/scala/kafka/server/LeaderEndPoint.scala: ## @@ -0,0 +1,66 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one or more + * contributor license agreements. See the NOTICE file distributed with + * this work for additional information regarding copyright ownership. + * The ASF licenses this file to You under the Apache License, Version 2.0 + * (the "License"); you may not use this file except in compliance with + * the License. You may obtain a copy of the License at + * + *http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ + +package kafka.server + +import kafka.api.{KAFKA_0_10_1_IV2, KAFKA_0_11_0_IV0, KAFKA_2_0_IV0, KAFKA_2_0_IV1, KAFKA_2_1_IV1, KAFKA_2_2_IV1, KAFKA_2_3_IV1, KAFKA_2_8_IV0, KAFKA_3_0_IV1} +import kafka.utils.Logging +import org.apache.kafka.common.TopicPartition +import org.apache.kafka.common.requests.FetchRequest +import org.apache.kafka.common.message.OffsetForLeaderEpochResponseData.EpochEndOffset +import org.apache.kafka.common.message.{FetchResponseData, OffsetForLeaderEpochRequestData} + +import scala.collection.Map + +trait LeaderEndPoint extends Logging { + val brokerConfig: KafkaConfig + val endpoint: BlockingSend = null + + type FetchData = FetchResponseData.PartitionData + type EpochData = OffsetForLeaderEpochRequestData.OffsetForLeaderPartition + + // Visible for testing + private[server] val listOffsetRequestVersion: Short = +if (brokerConfig.interBrokerProtocolVersion >= KAFKA_3_0_IV1) 7 +else if (brokerConfig.interBrokerProtocolVersion >= KAFKA_2_8_IV0) 6 +else if (brokerConfig.interBrokerProtocolVersion >= KAFKA_2_2_IV1) 5 +else if (brokerConfig.interBrokerProtocolVersion >= KAFKA_2_1_IV1) 4 +else if (brokerConfig.interBrokerProtocolVersion >= KAFKA_2_0_IV1) 3 +else if (brokerConfig.interBrokerProtocolVersion >= KAFKA_0_11_0_IV0) 2 +else if (brokerConfig.interBrokerProtocolVersion >= KAFKA_0_10_1_IV2) 1 +else 0 + + // Visible for testing + private[server] val offsetForLeaderEpochRequestVersion: Short = +if (brokerConfig.interBrokerProtocolVersion >= KAFKA_2_8_IV0) 4 +else if (brokerConfig.interBrokerProtocolVersion >= KAFKA_2_3_IV1) 3 +else if (brokerConfig.interBrokerProtocolVersion >= KAFKA_2_1_IV1) 2 +else if (brokerConfig.interBrokerProtocolVersion >= KAFKA_2_0_IV0) 1 +else 0 Review Comment: Do we really need those two in the trait as well? -- 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.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org
[GitHub] [kafka] dajac commented on a diff in pull request #12005: KAFKA-13803: Refactor Leader API Access
dajac commented on code in PR #12005: URL: https://github.com/apache/kafka/pull/12005#discussion_r845466434 ## core/src/main/scala/kafka/server/LeaderEndPoint.scala: ## @@ -0,0 +1,66 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one or more + * contributor license agreements. See the NOTICE file distributed with + * this work for additional information regarding copyright ownership. + * The ASF licenses this file to You under the Apache License, Version 2.0 + * (the "License"); you may not use this file except in compliance with + * the License. You may obtain a copy of the License at + * + *http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ + +package kafka.server + +import kafka.api.{KAFKA_0_10_1_IV2, KAFKA_0_11_0_IV0, KAFKA_2_0_IV0, KAFKA_2_0_IV1, KAFKA_2_1_IV1, KAFKA_2_2_IV1, KAFKA_2_3_IV1, KAFKA_2_8_IV0, KAFKA_3_0_IV1} +import kafka.utils.Logging +import org.apache.kafka.common.TopicPartition +import org.apache.kafka.common.requests.FetchRequest +import org.apache.kafka.common.message.OffsetForLeaderEpochResponseData.EpochEndOffset +import org.apache.kafka.common.message.{FetchResponseData, OffsetForLeaderEpochRequestData} + +import scala.collection.Map + +trait LeaderEndPoint extends Logging { + val brokerConfig: KafkaConfig + val endpoint: BlockingSend = null Review Comment: The `endpoint` seems to be only used by `RemoteLeaderEndPoint`. Could we remove it from the trait? -- 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.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org
[jira] [Commented] (KAFKA-13805) Upgrade vulnerable dependencies march 2022
[ https://issues.apache.org/jira/browse/KAFKA-13805?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17519093#comment-17519093 ] Bruno Cadonna commented on KAFKA-13805: --- [~kirktrue] Ah, got it! You are right! I updated the description. > Upgrade vulnerable dependencies march 2022 > -- > > Key: KAFKA-13805 > URL: https://issues.apache.org/jira/browse/KAFKA-13805 > Project: Kafka > Issue Type: Bug >Affects Versions: 2.8.1, 3.0.1 >Reporter: Shivakumar >Priority: Blocker > Labels: secutiry > Fix For: 2.8.2, 3.0.2 > > > https://nvd.nist.gov/vuln/detail/CVE-2020-36518 > |Packages|Package Version|CVSS|Fix Status| > |com.fasterxml.jackson.core_jackson-databind| 2.10.5.1| 7.5|fixed in 2.13.2.1| > |com.fasterxml.jackson.core_jackson-databind|2.13.1|7.5|fixed in 2.13.2.1| > Our security scan detected the above vulnerabilities > upgrade to correct versions for fixing vulnerabilities -- This message was sent by Atlassian Jira (v8.20.1#820001)
[jira] [Updated] (KAFKA-13805) Upgrade vulnerable dependencies march 2022
[ https://issues.apache.org/jira/browse/KAFKA-13805?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Bruno Cadonna updated KAFKA-13805: -- Description: https://nvd.nist.gov/vuln/detail/CVE-2020-36518 |Packages|Package Version|CVSS|Fix Status| |com.fasterxml.jackson.core_jackson-databind| 2.10.5.1| 7.5|fixed in 2.13.2.1| |com.fasterxml.jackson.core_jackson-databind|2.13.1|7.5|fixed in 2.13.2.1| Our security scan detected the above vulnerabilities upgrade to correct versions for fixing vulnerabilities was: |Packages|Package Version|CVSS|Fix Status| |com.fasterxml.jackson.core_jackson-databind| 2.10.5.1| 7.5|fixed in 2.13.0| |com.fasterxml.jackson.core_jackson-databind|2.13.1|7.5|fixed in 2.13.0| Our security scan detected the above vulnerabilities upgrade to correct versions for fixing vulnerabilities > Upgrade vulnerable dependencies march 2022 > -- > > Key: KAFKA-13805 > URL: https://issues.apache.org/jira/browse/KAFKA-13805 > Project: Kafka > Issue Type: Bug >Affects Versions: 2.8.1, 3.0.1 >Reporter: Shivakumar >Priority: Blocker > Labels: secutiry > Fix For: 2.8.2, 3.0.2 > > > https://nvd.nist.gov/vuln/detail/CVE-2020-36518 > |Packages|Package Version|CVSS|Fix Status| > |com.fasterxml.jackson.core_jackson-databind| 2.10.5.1| 7.5|fixed in 2.13.2.1| > |com.fasterxml.jackson.core_jackson-databind|2.13.1|7.5|fixed in 2.13.2.1| > Our security scan detected the above vulnerabilities > upgrade to correct versions for fixing vulnerabilities -- This message was sent by Atlassian Jira (v8.20.1#820001)
[GitHub] [kafka] xvrl commented on a diff in pull request #11998: KAFKA-13801: Kafka server does not respect MetricsReporter contract for dynamically configured reporters
xvrl commented on code in PR #11998: URL: https://github.com/apache/kafka/pull/11998#discussion_r845459993 ## core/src/test/scala/integration/kafka/server/DynamicBrokerReconfigurationTest.scala: ## @@ -1813,14 +1812,20 @@ class TestMetricsReporter extends MetricsReporter with Reconfigurable with Close import TestMetricsReporter._ val kafkaMetrics = ArrayBuffer[KafkaMetric]() @volatile var initializeCount = 0 + @volatile var contextChangeCount = 0 @volatile var configureCount = 0 @volatile var reconfigureCount = 0 @volatile var closeCount = 0 @volatile var clusterUpdateCount = 0 @volatile var pollingInterval: Int = -1 testReporters.add(this) + override def contextChange(metricsContext: MetricsContext): Unit = { +contextChangeCount += 1 + } + override def init(metrics: util.List[KafkaMetric]): Unit = { +assertTrue(contextChangeCount > 0, "contextChange must be called before init") Review Comment: we technically don't specify exactly how many times it gets called before, but in practice it's once yes. We could certainly evolve the contract and be more explicit -- 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.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org
[GitHub] [kafka] cadonna opened a new pull request, #12013: Reenable flaky tests
cadonna opened a new pull request, #12013: URL: https://github.com/apache/kafka/pull/12013 Just a test ### Committer Checklist (excluded from commit message) - [ ] Verify design and implementation - [ ] Verify test coverage and CI build status - [ ] Verify documentation (including upgrade notes) -- 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.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org
[jira] [Comment Edited] (KAFKA-13805) Upgrade vulnerable dependencies march 2022
[ https://issues.apache.org/jira/browse/KAFKA-13805?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17519087#comment-17519087 ] Kirk True edited comment on KAFKA-13805 at 4/7/22 6:37 PM: --- [~cadonna] - Sorry for the confusion... I was mentioning the 2.13.x line because the description stated that the issue was "fixed in 2.13.0", which I don't believe is accurate. was (Author: kirktrue): [~cadonna] - Sorry for the confusion... I was mentioning the 2.13.x line because the description stated that the issue was "fixed in 2.13.0", which I don't believe is accurate. Also: {quote}This CVE seems to affect 2.8.1, 3.0.1 but not 3.1.1 and 3.2.0 since the latter ones use 2.12.6.1 (see KAFKA-13658). {quote} When I look at [{{dependencies.gradle}} in trunk|https://github.com/apache/kafka/blob/trunk/gradle/dependencies.gradle#L69], {{2.12.6}} not {{2.12.6.1}} is listed as the version of Jackson libraries used. So {{3.1.1}} and {{3.2.0}} are affected too, right? > Upgrade vulnerable dependencies march 2022 > -- > > Key: KAFKA-13805 > URL: https://issues.apache.org/jira/browse/KAFKA-13805 > Project: Kafka > Issue Type: Bug >Affects Versions: 2.8.1, 3.0.1 >Reporter: Shivakumar >Priority: Blocker > Labels: secutiry > Fix For: 2.8.2, 3.0.2 > > > |Packages|Package Version|CVSS|Fix Status| > |com.fasterxml.jackson.core_jackson-databind| 2.10.5.1| 7.5|fixed in 2.13.0| > |com.fasterxml.jackson.core_jackson-databind|2.13.1|7.5|fixed in 2.13.0| > Our security scan detected the above vulnerabilities > upgrade to correct versions for fixing vulnerabilities -- This message was sent by Atlassian Jira (v8.20.1#820001)
[jira] [Commented] (KAFKA-13805) Upgrade vulnerable dependencies march 2022
[ https://issues.apache.org/jira/browse/KAFKA-13805?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17519087#comment-17519087 ] Kirk True commented on KAFKA-13805: --- [~cadonna] - Sorry for the confusion... I was mentioning the 2.13.x line because the description stated that the issue was "fixed in 2.13.0", which I don't believe is accurate. Also: {quote}This CVE seems to affect 2.8.1, 3.0.1 but not 3.1.1 and 3.2.0 since the latter ones use 2.12.6.1 (see KAFKA-13658). {quote} When I look at [{{dependencies.gradle}} in trunk|https://github.com/apache/kafka/blob/trunk/gradle/dependencies.gradle#L69], {{2.12.6}} not {{2.12.6.1}} is listed as the version of Jackson libraries used. So {{3.1.1}} and {{3.2.0}} are affected too, right? > Upgrade vulnerable dependencies march 2022 > -- > > Key: KAFKA-13805 > URL: https://issues.apache.org/jira/browse/KAFKA-13805 > Project: Kafka > Issue Type: Bug >Affects Versions: 2.8.1, 3.0.1 >Reporter: Shivakumar >Priority: Blocker > Labels: secutiry > Fix For: 2.8.2, 3.0.2 > > > |Packages|Package Version|CVSS|Fix Status| > |com.fasterxml.jackson.core_jackson-databind| 2.10.5.1| 7.5|fixed in 2.13.0| > |com.fasterxml.jackson.core_jackson-databind|2.13.1|7.5|fixed in 2.13.0| > Our security scan detected the above vulnerabilities > upgrade to correct versions for fixing vulnerabilities -- This message was sent by Atlassian Jira (v8.20.1#820001)
[jira] [Commented] (KAFKA-13805) Upgrade vulnerable dependencies march 2022
[ https://issues.apache.org/jira/browse/KAFKA-13805?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17519082#comment-17519082 ] Bruno Cadonna commented on KAFKA-13805: --- [~kirktrue] Under "Known Affected Software Configurations" the CVE says "Up to (excluding) 2.12.6.1". We are not using the 2.13.x line. > Upgrade vulnerable dependencies march 2022 > -- > > Key: KAFKA-13805 > URL: https://issues.apache.org/jira/browse/KAFKA-13805 > Project: Kafka > Issue Type: Bug >Affects Versions: 2.8.1, 3.0.1 >Reporter: Shivakumar >Priority: Blocker > Labels: secutiry > Fix For: 2.8.2, 3.0.2 > > > |Packages|Package Version|CVSS|Fix Status| > |com.fasterxml.jackson.core_jackson-databind| 2.10.5.1| 7.5|fixed in 2.13.0| > |com.fasterxml.jackson.core_jackson-databind|2.13.1|7.5|fixed in 2.13.0| > Our security scan detected the above vulnerabilities > upgrade to correct versions for fixing vulnerabilities -- This message was sent by Atlassian Jira (v8.20.1#820001)
[jira] [Created] (KAFKA-13806) Check CRC when reading snapshots
Jose Armando Garcia Sancio created KAFKA-13806: -- Summary: Check CRC when reading snapshots Key: KAFKA-13806 URL: https://issues.apache.org/jira/browse/KAFKA-13806 Project: Kafka Issue Type: Sub-task Reporter: Jose Armando Garcia Sancio Assignee: Jose Armando Garcia Sancio -- This message was sent by Atlassian Jira (v8.20.1#820001)
[GitHub] [kafka] kirktrue commented on a diff in pull request #11998: KAFKA-13801: Kafka server does not respect MetricsReporter contract for dynamically configured reporters
kirktrue commented on code in PR #11998: URL: https://github.com/apache/kafka/pull/11998#discussion_r845436714 ## core/src/test/scala/integration/kafka/server/DynamicBrokerReconfigurationTest.scala: ## @@ -1813,14 +1812,20 @@ class TestMetricsReporter extends MetricsReporter with Reconfigurable with Close import TestMetricsReporter._ val kafkaMetrics = ArrayBuffer[KafkaMetric]() @volatile var initializeCount = 0 + @volatile var contextChangeCount = 0 @volatile var configureCount = 0 @volatile var reconfigureCount = 0 @volatile var closeCount = 0 @volatile var clusterUpdateCount = 0 @volatile var pollingInterval: Int = -1 testReporters.add(this) + override def contextChange(metricsContext: MetricsContext): Unit = { +contextChangeCount += 1 + } + override def init(metrics: util.List[KafkaMetric]): Unit = { +assertTrue(contextChangeCount > 0, "contextChange must be called before init") Review Comment: I wonder if it would be safer to test `contextChangeCount == 1` so that we know it's been called once and only once when `init` is called? According to the JavaDoc, `contextChange` "may be called anytime after" `init` is called, but only once beforehand, right? -- 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.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org
[GitHub] [kafka] artemlivshits commented on a diff in pull request #12006: KAFKA-13794: Follow up to fix comparator
artemlivshits commented on code in PR #12006: URL: https://github.com/apache/kafka/pull/12006#discussion_r845433136 ## clients/src/main/java/org/apache/kafka/clients/producer/internals/TransactionManager.java: ## @@ -187,7 +187,7 @@ private void startSequencesAtBeginning(TopicPartition topicPartition, ProducerId private static final Comparator PRODUCER_BATCH_COMPARATOR = (b1, b2) -> { if (b1.baseSequence() < b2.baseSequence()) return -1; else if (b1.baseSequence() > b2.baseSequence()) return 1; -else return b1.equals(b2) ? 0 : 1; +else return b1.equals(b2) ? 0 : Integer.compare(b1.hashCode(), b2.hashCode()); Review Comment: `ProducerBatch` doesn't override hashCode, so a default implementation is used. It's not fully specified what the default implementation of hashCode has to return, but it looks like the suggested implementation is to return a value based on the object address, so it should be different for different objects. BTW, we could probably just do ``` else return Integer.compare(b1.hashCode(), b2.hashCode()); ``` if the objects are equal, their hash code must be equal too. -- 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.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org
[jira] [Comment Edited] (KAFKA-13805) Upgrade vulnerable dependencies march 2022
[ https://issues.apache.org/jira/browse/KAFKA-13805?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17519073#comment-17519073 ] Kirk True edited comment on KAFKA-13805 at 4/7/22 6:07 PM: --- According to [https://mvnrepository.com/artifact/com.fasterxml.jackson.core/jackson-databind,] 2.13.0 still has the vulnerability. 2.13.2.1 looks to be the first version in the 2.13.x line that has the fix. was (Author: kirktrue): According to [https://mvnrepository.com/artifact/com.fasterxml.jackson.core/jackson-databind,] 2.13.0 still has the vulnerability. 2.13.2.1 looks to be the first version in the 2.13. line that has the fix. > Upgrade vulnerable dependencies march 2022 > -- > > Key: KAFKA-13805 > URL: https://issues.apache.org/jira/browse/KAFKA-13805 > Project: Kafka > Issue Type: Bug >Affects Versions: 2.8.1, 3.0.1 >Reporter: Shivakumar >Priority: Blocker > Labels: secutiry > Fix For: 2.8.2, 3.0.2 > > > |Packages|Package Version|CVSS|Fix Status| > |com.fasterxml.jackson.core_jackson-databind| 2.10.5.1| 7.5|fixed in 2.13.0| > |com.fasterxml.jackson.core_jackson-databind|2.13.1|7.5|fixed in 2.13.0| > Our security scan detected the above vulnerabilities > upgrade to correct versions for fixing vulnerabilities -- This message was sent by Atlassian Jira (v8.20.1#820001)
[jira] [Commented] (KAFKA-13805) Upgrade vulnerable dependencies march 2022
[ https://issues.apache.org/jira/browse/KAFKA-13805?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17519073#comment-17519073 ] Kirk True commented on KAFKA-13805: --- According to [https://mvnrepository.com/artifact/com.fasterxml.jackson.core/jackson-databind,] 2.13.0 still has the vulnerability. 2.13.2.1 looks to be the first version in the 2.13. line that has the fix. > Upgrade vulnerable dependencies march 2022 > -- > > Key: KAFKA-13805 > URL: https://issues.apache.org/jira/browse/KAFKA-13805 > Project: Kafka > Issue Type: Bug >Affects Versions: 2.8.1, 3.0.1 >Reporter: Shivakumar >Priority: Blocker > Labels: secutiry > Fix For: 2.8.2, 3.0.2 > > > |Packages|Package Version|CVSS|Fix Status| > |com.fasterxml.jackson.core_jackson-databind| 2.10.5.1| 7.5|fixed in 2.13.0| > |com.fasterxml.jackson.core_jackson-databind|2.13.1|7.5|fixed in 2.13.0| > Our security scan detected the above vulnerabilities > upgrade to correct versions for fixing vulnerabilities -- This message was sent by Atlassian Jira (v8.20.1#820001)
[jira] [Updated] (KAFKA-13805) Upgrade vulnerable dependencies march 2022
[ https://issues.apache.org/jira/browse/KAFKA-13805?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Bruno Cadonna updated KAFKA-13805: -- Fix Version/s: (was: 3.1.2) > Upgrade vulnerable dependencies march 2022 > -- > > Key: KAFKA-13805 > URL: https://issues.apache.org/jira/browse/KAFKA-13805 > Project: Kafka > Issue Type: Bug >Affects Versions: 2.8.1, 3.0.1, 3.1.1 >Reporter: Shivakumar >Priority: Blocker > Labels: secutiry > Fix For: 2.8.2, 3.0.2 > > > |Packages|Package Version|CVSS|Fix Status| > |com.fasterxml.jackson.core_jackson-databind| 2.10.5.1| 7.5|fixed in 2.13.0| > |com.fasterxml.jackson.core_jackson-databind|2.13.1|7.5|fixed in 2.13.0| > Our security scan detected the above vulnerabilities > upgrade to correct versions for fixing vulnerabilities -- This message was sent by Atlassian Jira (v8.20.1#820001)
[jira] [Updated] (KAFKA-13805) Upgrade vulnerable dependencies march 2022
[ https://issues.apache.org/jira/browse/KAFKA-13805?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Bruno Cadonna updated KAFKA-13805: -- Affects Version/s: (was: 3.1.1) > Upgrade vulnerable dependencies march 2022 > -- > > Key: KAFKA-13805 > URL: https://issues.apache.org/jira/browse/KAFKA-13805 > Project: Kafka > Issue Type: Bug >Affects Versions: 2.8.1, 3.0.1 >Reporter: Shivakumar >Priority: Blocker > Labels: secutiry > Fix For: 2.8.2, 3.0.2 > > > |Packages|Package Version|CVSS|Fix Status| > |com.fasterxml.jackson.core_jackson-databind| 2.10.5.1| 7.5|fixed in 2.13.0| > |com.fasterxml.jackson.core_jackson-databind|2.13.1|7.5|fixed in 2.13.0| > Our security scan detected the above vulnerabilities > upgrade to correct versions for fixing vulnerabilities -- This message was sent by Atlassian Jira (v8.20.1#820001)
[jira] [Comment Edited] (KAFKA-13805) Upgrade vulnerable dependencies march 2022
[ https://issues.apache.org/jira/browse/KAFKA-13805?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17519059#comment-17519059 ] Bruno Cadonna edited comment on KAFKA-13805 at 4/7/22 5:45 PM: --- [~shivakumar] Are you referring to the following CVE? https://nvd.nist.gov/vuln/detail/CVE-2020-36518 This CVE seems to affect 2.8.1, 3.0.1 but not 3.1.1 and 3.2.0 since the latter ones use 2.12.6.1 (see KAFKA-13658). was (Author: cadonna): [~shivakumar] Are you referring to the following CVE? https://nvd.nist.gov/vuln/detail/CVE-2020-36518 This CVE seems to affect 2.8.1, 3.0.1, and 3.1.1 (\cc [~tombentley]) but not 3.2.0 since 3.2.0 uses 2.12.6.1 (see KAFKA-13658). > Upgrade vulnerable dependencies march 2022 > -- > > Key: KAFKA-13805 > URL: https://issues.apache.org/jira/browse/KAFKA-13805 > Project: Kafka > Issue Type: Bug >Affects Versions: 2.8.1, 3.0.1, 3.1.1 >Reporter: Shivakumar >Priority: Blocker > Labels: secutiry > Fix For: 2.8.2, 3.0.2, 3.1.2 > > > |Packages|Package Version|CVSS|Fix Status| > |com.fasterxml.jackson.core_jackson-databind| 2.10.5.1| 7.5|fixed in 2.13.0| > |com.fasterxml.jackson.core_jackson-databind|2.13.1|7.5|fixed in 2.13.0| > Our security scan detected the above vulnerabilities > upgrade to correct versions for fixing vulnerabilities -- This message was sent by Atlassian Jira (v8.20.1#820001)
[jira] [Commented] (KAFKA-13805) Upgrade vulnerable dependencies march 2022
[ https://issues.apache.org/jira/browse/KAFKA-13805?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17519059#comment-17519059 ] Bruno Cadonna commented on KAFKA-13805: --- [~shivakumar] Are you referring to the following CVE? https://nvd.nist.gov/vuln/detail/CVE-2020-36518 This CVE seems to affect 2.8.1, 3.0.1, and 3.1.1 (\cc [~tombentley]) but not 3.2.0 since 3.2.0 uses 2.12.6.1 (see KAFKA-13658). > Upgrade vulnerable dependencies march 2022 > -- > > Key: KAFKA-13805 > URL: https://issues.apache.org/jira/browse/KAFKA-13805 > Project: Kafka > Issue Type: Bug >Affects Versions: 2.8.1, 3.0.1, 3.1.1 >Reporter: Shivakumar >Priority: Blocker > Labels: secutiry > Fix For: 2.8.2, 3.0.2, 3.1.2 > > > |Packages|Package Version|CVSS|Fix Status| > |com.fasterxml.jackson.core_jackson-databind| 2.10.5.1| 7.5|fixed in 2.13.0| > |com.fasterxml.jackson.core_jackson-databind|2.13.1|7.5|fixed in 2.13.0| > Our security scan detected the above vulnerabilities > upgrade to correct versions for fixing vulnerabilities -- This message was sent by Atlassian Jira (v8.20.1#820001)
[jira] [Updated] (KAFKA-13805) Upgrade vulnerable dependencies march 2022
[ https://issues.apache.org/jira/browse/KAFKA-13805?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Bruno Cadonna updated KAFKA-13805: -- Fix Version/s: 3.1.2 2.8.2 3.0.2 > Upgrade vulnerable dependencies march 2022 > -- > > Key: KAFKA-13805 > URL: https://issues.apache.org/jira/browse/KAFKA-13805 > Project: Kafka > Issue Type: Bug >Affects Versions: 2.8.1, 3.0.1, 3.1.1 >Reporter: Shivakumar >Priority: Blocker > Labels: secutiry > Fix For: 2.8.2, 3.0.2, 3.1.2 > > > |Packages|Package Version|CVSS|Fix Status| > |com.fasterxml.jackson.core_jackson-databind| 2.10.5.1| 7.5|fixed in 2.13.0| > |com.fasterxml.jackson.core_jackson-databind|2.13.1|7.5|fixed in 2.13.0| > Our security scan detected the above vulnerabilities > upgrade to correct versions for fixing vulnerabilities -- This message was sent by Atlassian Jira (v8.20.1#820001)
[jira] [Updated] (KAFKA-13805) Upgrade vulnerable dependencies march 2022
[ https://issues.apache.org/jira/browse/KAFKA-13805?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Bruno Cadonna updated KAFKA-13805: -- Priority: Blocker (was: Major) > Upgrade vulnerable dependencies march 2022 > -- > > Key: KAFKA-13805 > URL: https://issues.apache.org/jira/browse/KAFKA-13805 > Project: Kafka > Issue Type: Bug >Affects Versions: 2.8.1, 3.0.1, 3.1.1 >Reporter: Shivakumar >Priority: Blocker > Labels: secutiry > Fix For: 3.0.1, 3.2.0, 3.1.1 > > > |Packages|Package Version|CVSS|Fix Status| > |com.fasterxml.jackson.core_jackson-databind| 2.10.5.1| 7.5|fixed in 2.13.0| > |com.fasterxml.jackson.core_jackson-databind|2.13.1|7.5|fixed in 2.13.0| > Our security scan detected the above vulnerabilities > upgrade to correct versions for fixing vulnerabilities -- This message was sent by Atlassian Jira (v8.20.1#820001)
[jira] [Updated] (KAFKA-13805) Upgrade vulnerable dependencies march 2022
[ https://issues.apache.org/jira/browse/KAFKA-13805?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Bruno Cadonna updated KAFKA-13805: -- Fix Version/s: (was: 3.0.1) (was: 3.2.0) (was: 3.1.1) > Upgrade vulnerable dependencies march 2022 > -- > > Key: KAFKA-13805 > URL: https://issues.apache.org/jira/browse/KAFKA-13805 > Project: Kafka > Issue Type: Bug >Affects Versions: 2.8.1, 3.0.1, 3.1.1 >Reporter: Shivakumar >Priority: Blocker > Labels: secutiry > > |Packages|Package Version|CVSS|Fix Status| > |com.fasterxml.jackson.core_jackson-databind| 2.10.5.1| 7.5|fixed in 2.13.0| > |com.fasterxml.jackson.core_jackson-databind|2.13.1|7.5|fixed in 2.13.0| > Our security scan detected the above vulnerabilities > upgrade to correct versions for fixing vulnerabilities -- This message was sent by Atlassian Jira (v8.20.1#820001)
[jira] [Updated] (KAFKA-13805) Upgrade vulnerable dependencies march 2022
[ https://issues.apache.org/jira/browse/KAFKA-13805?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Bruno Cadonna updated KAFKA-13805: -- Affects Version/s: 3.0.1 3.1.1 > Upgrade vulnerable dependencies march 2022 > -- > > Key: KAFKA-13805 > URL: https://issues.apache.org/jira/browse/KAFKA-13805 > Project: Kafka > Issue Type: Bug >Affects Versions: 2.8.1, 3.0.1, 3.1.1 >Reporter: Shivakumar >Priority: Major > Labels: secutiry > Fix For: 3.0.1, 3.2.0, 3.1.1 > > > |Packages|Package Version|CVSS|Fix Status| > |com.fasterxml.jackson.core_jackson-databind| 2.10.5.1| 7.5|fixed in 2.13.0| > |com.fasterxml.jackson.core_jackson-databind|2.13.1|7.5|fixed in 2.13.0| > Our security scan detected the above vulnerabilities > upgrade to correct versions for fixing vulnerabilities -- This message was sent by Atlassian Jira (v8.20.1#820001)
[jira] [Updated] (KAFKA-8575) Investigate removing EAGER protocol & cleaning up task suspension in Streams rebalancing
[ https://issues.apache.org/jira/browse/KAFKA-8575?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Bruno Cadonna updated KAFKA-8575: - Priority: Critical (was: Blocker) > Investigate removing EAGER protocol & cleaning up task suspension in Streams > rebalancing > - > > Key: KAFKA-8575 > URL: https://issues.apache.org/jira/browse/KAFKA-8575 > Project: Kafka > Issue Type: Improvement > Components: streams >Affects Versions: 2.4.0 >Reporter: A. Sophie Blee-Goldman >Priority: Critical > Fix For: 3.2.0 > > > With KIP-429 the suspend/resume of tasks may have minimal gains while adding > a lot of complexity and potential bugs. We should consider removing/cleaning > it up and going a step further to remove the EAGER protocol from Streams > entirely. > Plan to remove this in 3.1/4.0, whichever comes after 3.0. This will make 3.0 > a bridge release for users upgrading from any version below 2.4, but they > will still be able to do so in the usual two rolling bounces. > > *The upgrade path from 2.3 and below, to any \{to_version} higher than 3.1 > will be:* > 1. During the first rolling bounce, upgrade the jars to a version between 2.4 > - 3.1 and add the UPGRADE_FROM config for whichever version you are upgrading > from > 2. During the second rolling bounce, upgrade the jars to the desired > \{to_version} and remove the UPGRADE_FROM config > > EAGER will be effectively deprecated in 3.0 but not removed until the next > version. -- This message was sent by Atlassian Jira (v8.20.1#820001)
[jira] [Updated] (KAFKA-8575) Investigate removing EAGER protocol & cleaning up task suspension in Streams rebalancing
[ https://issues.apache.org/jira/browse/KAFKA-8575?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Bruno Cadonna updated KAFKA-8575: - Fix Version/s: (was: 3.2.0) > Investigate removing EAGER protocol & cleaning up task suspension in Streams > rebalancing > - > > Key: KAFKA-8575 > URL: https://issues.apache.org/jira/browse/KAFKA-8575 > Project: Kafka > Issue Type: Improvement > Components: streams >Affects Versions: 2.4.0 >Reporter: A. Sophie Blee-Goldman >Priority: Critical > > With KIP-429 the suspend/resume of tasks may have minimal gains while adding > a lot of complexity and potential bugs. We should consider removing/cleaning > it up and going a step further to remove the EAGER protocol from Streams > entirely. > Plan to remove this in 3.1/4.0, whichever comes after 3.0. This will make 3.0 > a bridge release for users upgrading from any version below 2.4, but they > will still be able to do so in the usual two rolling bounces. > > *The upgrade path from 2.3 and below, to any \{to_version} higher than 3.1 > will be:* > 1. During the first rolling bounce, upgrade the jars to a version between 2.4 > - 3.1 and add the UPGRADE_FROM config for whichever version you are upgrading > from > 2. During the second rolling bounce, upgrade the jars to the desired > \{to_version} and remove the UPGRADE_FROM config > > EAGER will be effectively deprecated in 3.0 but not removed until the next > version. -- This message was sent by Atlassian Jira (v8.20.1#820001)
[jira] [Updated] (KAFKA-6204) Interceptor and MetricsReporter should implement java.io.Closeable
[ https://issues.apache.org/jira/browse/KAFKA-6204?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Xavier Léauté updated KAFKA-6204: - Fix Version/s: 3.3.0 (was: 3.2.0) > Interceptor and MetricsReporter should implement java.io.Closeable > -- > > Key: KAFKA-6204 > URL: https://issues.apache.org/jira/browse/KAFKA-6204 > Project: Kafka > Issue Type: Improvement > Components: clients >Reporter: Charly Molter >Assignee: Xavier Léauté >Priority: Minor > Fix For: 3.3.0 > > > The serializers and deserializers extends the Closeable interface, even > ConsumerInterceptors and ProducerInterceptors implement it. > ConsumerInterceptor, ProducerInterceptor and MetricsReporter do not extend > the Closeable interface. > Maybe they should for coherency with the rest of the apis. -- This message was sent by Atlassian Jira (v8.20.1#820001)
[jira] [Updated] (KAFKA-13805) Upgrade vulnerable dependencies march 2022
[ https://issues.apache.org/jira/browse/KAFKA-13805?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Shivakumar updated KAFKA-13805: --- Description: |Packages|Package Version|CVSS|Fix Status| |com.fasterxml.jackson.core_jackson-databind| 2.10.5.1| 7.5|fixed in 2.13.0| |com.fasterxml.jackson.core_jackson-databind|2.13.1|7.5|fixed in 2.13.0| Our security scan detected the above vulnerabilities upgrade to correct versions for fixing vulnerabilities was: |Packages|Package Version|CVSS|Fix Status| |com.fasterxml.jackson.core_jackson-databind| 2.10.5.1| 7.5| fixed in 2.14, 2.13.1, 2.12.6| | | | | | Our security scan detected the above vulnerabilities upgrade to correct versions for fixing vulnerabilities > Upgrade vulnerable dependencies march 2022 > -- > > Key: KAFKA-13805 > URL: https://issues.apache.org/jira/browse/KAFKA-13805 > Project: Kafka > Issue Type: Bug >Affects Versions: 2.8.1 >Reporter: Shivakumar >Priority: Major > Labels: secutiry > Fix For: 3.0.1, 3.2.0, 3.1.1 > > > |Packages|Package Version|CVSS|Fix Status| > |com.fasterxml.jackson.core_jackson-databind| 2.10.5.1| 7.5|fixed in 2.13.0| > |com.fasterxml.jackson.core_jackson-databind|2.13.1|7.5|fixed in 2.13.0| > Our security scan detected the above vulnerabilities > upgrade to correct versions for fixing vulnerabilities -- This message was sent by Atlassian Jira (v8.20.1#820001)
[jira] [Created] (KAFKA-13805) Upgrade vulnerable dependencies march 2022
Shivakumar created KAFKA-13805: -- Summary: Upgrade vulnerable dependencies march 2022 Key: KAFKA-13805 URL: https://issues.apache.org/jira/browse/KAFKA-13805 Project: Kafka Issue Type: Bug Affects Versions: 2.8.1 Reporter: Shivakumar Fix For: 3.0.1, 3.2.0, 3.1.1 |Packages|Package Version|CVSS|Fix Status| |com.fasterxml.jackson.core_jackson-databind| 2.10.5.1| 7.5| fixed in 2.14, 2.13.1, 2.12.6| | | | | | Our security scan detected the above vulnerabilities upgrade to correct versions for fixing vulnerabilities -- This message was sent by Atlassian Jira (v8.20.1#820001)
[GitHub] [kafka] ddrid commented on a diff in pull request #12006: KAFKA-13794: Follow up to fix comparator
ddrid commented on code in PR #12006: URL: https://github.com/apache/kafka/pull/12006#discussion_r844881404 ## clients/src/main/java/org/apache/kafka/clients/producer/internals/TransactionManager.java: ## @@ -187,7 +187,7 @@ private void startSequencesAtBeginning(TopicPartition topicPartition, ProducerId private static final Comparator PRODUCER_BATCH_COMPARATOR = (b1, b2) -> { if (b1.baseSequence() < b2.baseSequence()) return -1; else if (b1.baseSequence() > b2.baseSequence()) return 1; -else return b1.equals(b2) ? 0 : 1; +else return b1.equals(b2) ? 0 : Integer.compare(b1.hashCode(), b2.hashCode()); Review Comment: Hi @showuon, thanks for you comment. I found some trouble when adding `equals` method in `ProducerBatch`. The only two final fields I can use is `createdMs` and `topicPartition`, I don't think it make sense if I use these two fields since they usually have duplicates. Do you have any suggestions? Thanks -- 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.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org
[jira] [Comment Edited] (KAFKA-13571) Enabling MirrorMaker 2.0 with TLS
[ https://issues.apache.org/jira/browse/KAFKA-13571?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17518846#comment-17518846 ] Jordan Moore edited comment on KAFKA-13571 at 4/7/22 12:21 PM: --- Please lower the priority as I don't believe this is a project blocker. It should work just fine. Please check the startup logs for the process to verify the consumer and producer configs have loaded the properties you've defined. was (Author: cricket007): It should work just fine. Please check the startup logs for the process to verify the consumer and producer configs have loaded the properties you've defined. > Enabling MirrorMaker 2.0 with TLS > - > > Key: KAFKA-13571 > URL: https://issues.apache.org/jira/browse/KAFKA-13571 > Project: Kafka > Issue Type: Task > Components: mirrormaker >Affects Versions: 3.0.0 >Reporter: Bharath Reddy >Priority: Blocker > > Hi All, > > We are trying to enableTLS for MirrorMaker 2.0(connect-mirror-maker.sh) for > apache kafka 3.0.I have used below parameters but it has not succeeded. > > Please confirm the points below. > > - TLS feature is available for MirrorMaker 2.0(connect-mirror-maker.sh),If > yes can you please share a blog/configuration to enable it. > > source.ssl.truststore.location=/home/kafka.truststore.jks > source.ssl.truststore.password= > source.ssl.keystore.location=/home/kafka.keystore.jks > source.ssl.keystore.password=** > source.ssl.key.password=** > source.security.inter.broker.protocol=SSL > source.ssl.endpoint.identification.algorithm= > target.ssl.truststore.location=/home//kafka.truststore.jks > target.ssl.truststore.password=*** > target.ssl.keystore.location=/home/kafka.keystore.jks > target.ssl.keystore.password=** > target.ssl.key.password=** > target.security.inter.broker.protocol=SSL > target.ssl.endpoint.identification.algorithm= > > Thanks, > Bharath Reddy -- This message was sent by Atlassian Jira (v8.20.1#820001)
[jira] [Commented] (KAFKA-13571) Enabling MirrorMaker 2.0 with TLS
[ https://issues.apache.org/jira/browse/KAFKA-13571?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17518846#comment-17518846 ] Jordan Moore commented on KAFKA-13571: -- It should work just fine. Please check the startup logs for the process to verify the consumer and producer configs have loaded the properties you've defined. > Enabling MirrorMaker 2.0 with TLS > - > > Key: KAFKA-13571 > URL: https://issues.apache.org/jira/browse/KAFKA-13571 > Project: Kafka > Issue Type: Task > Components: mirrormaker >Affects Versions: 3.0.0 >Reporter: Bharath Reddy >Priority: Blocker > > Hi All, > > We are trying to enableTLS for MirrorMaker 2.0(connect-mirror-maker.sh) for > apache kafka 3.0.I have used below parameters but it has not succeeded. > > Please confirm the points below. > > - TLS feature is available for MirrorMaker 2.0(connect-mirror-maker.sh),If > yes can you please share a blog/configuration to enable it. > > source.ssl.truststore.location=/home/kafka.truststore.jks > source.ssl.truststore.password= > source.ssl.keystore.location=/home/kafka.keystore.jks > source.ssl.keystore.password=** > source.ssl.key.password=** > source.security.inter.broker.protocol=SSL > source.ssl.endpoint.identification.algorithm= > target.ssl.truststore.location=/home//kafka.truststore.jks > target.ssl.truststore.password=*** > target.ssl.keystore.location=/home/kafka.keystore.jks > target.ssl.keystore.password=** > target.ssl.key.password=** > target.security.inter.broker.protocol=SSL > target.ssl.endpoint.identification.algorithm= > > Thanks, > Bharath Reddy -- This message was sent by Atlassian Jira (v8.20.1#820001)
[jira] [Commented] (KAFKA-13762) Kafka brokers are not coming up
[ https://issues.apache.org/jira/browse/KAFKA-13762?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17518845#comment-17518845 ] Jordan Moore commented on KAFKA-13762: -- [~kkameshm90] Please lower the priority as it is not a project blocker. Your error is caused by the JMX exporter from Prometheus you've custom added to your broker startup process, and so is really unrelated to any issues with Kafka project. You need to stop whatever other process has already bound to the exporter port or change the port the exporter runs on > Kafka brokers are not coming up > > > Key: KAFKA-13762 > URL: https://issues.apache.org/jira/browse/KAFKA-13762 > Project: Kafka > Issue Type: Bug >Reporter: Kamesh >Priority: Blocker > > Out of 9 brokers only 3 brokers coming up. Totally 3 VMs Each VM is having 3 > brokers > We are getting below error > Exception in thread "main" java.lang.reflect.InvocationTargetException > at sun.reflect.NativeMethodAccessorImpl.invoke0(Native Method) > at > sun.reflect.NativeMethodAccessorImpl.invoke(NativeMethodAccessorImpl.java:62) > at > sun.reflect.DelegatingMethodAccessorImpl.invoke(DelegatingMethodAccessorImpl.java:43) > at java.lang.reflect.Method.invoke(Method.java:498) > at > sun.instrument.InstrumentationImpl.loadClassAndStartAgent(InstrumentationImpl.java:386) > at > sun.instrument.InstrumentationImpl.loadClassAndCallPremain(InstrumentationImpl.java:401) > Caused by: java.net.BindException: Address already in use > at sun.nio.ch.Net.bind0(Native Method) > at sun.nio.ch.Net.bind(Net.java:433) > at sun.nio.ch.Net.bind(Net.java:425) > at > sun.nio.ch.ServerSocketChannelImpl.bind(ServerSocketChannelImpl.java:223) > at sun.nio.ch.ServerSocketAdaptor.bind(ServerSocketAdaptor.java:74) > at sun.net.httpserver.ServerImpl.(ServerImpl.java:100) > at sun.net.httpserver.HttpServerImpl.(HttpServerImpl.java:50) > at > sun.net.httpserver.DefaultHttpServerProvider.createHttpServer(DefaultHttpServerProvider.java:35) > at com.sun.net.httpserver.HttpServer.create(HttpServer.java:130) > at > io.prometheus.jmx.shaded.io.prometheus.client.exporter.HTTPServer.(HTTPServer.java:179) > at > io.prometheus.jmx.shaded.io.prometheus.jmx.JavaAgent.premain(JavaAgent.java:31) -- This message was sent by Atlassian Jira (v8.20.1#820001)
[GitHub] [kafka] EvansJahja opened a new pull request, #12012: MINOR: add print.topic property in console consumer
EvansJahja opened a new pull request, #12012: URL: https://github.com/apache/kafka/pull/12012 ``` Usage: ./kafka-console-consumer.sh --bootstrap-server kafka:9092 --include '.*' --property print.topic=true Example output: Topic:hello.world{"msg": "abc"} ``` Purpose: Until now, there is no way to print all messages from all topics while maintaining the topic information. This PR adds this functionallity. -- 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.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org
[GitHub] [kafka] EvansJahja closed pull request #12011: MINOR: add print.topic property in console consumer
EvansJahja closed pull request #12011: MINOR: add print.topic property in console consumer URL: https://github.com/apache/kafka/pull/12011 -- 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.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org
[jira] [Updated] (KAFKA-13804) Log broker shutdown reason at the end of log output
[ https://issues.apache.org/jira/browse/KAFKA-13804?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Luke Chen updated KAFKA-13804: -- Description: Currently, when the broker hit exception, or SIGINT signal received, we will first log the exception (or the reason we started to shutdown) ex: _INFO Terminating process due to signal SIGINT_ And then go through the shutdown process, to close all the resources (i.e. threads, sockets, metrics...), and in the end, output: _[KafkaServer id=0] shut down completed (kafka.server.KafkaServer)_ Sometimes, during closing the resources, we will also hit some exceptions, or waiting for some other dependencies close, and output many more logs. When we tried to find out why the broker is shutdown from the log, we have to go through many lines of logs from the tail, and skip some unrelated exceptions, and then identify the specific reason why the broker shutting down. We could improve it by adding the reason why the broker is shutting down at the end of log, something like this: {code:java} [2022-04-07 18:19:33,005] INFO Terminating process due to signal SIGINT (org.apache.kafka.common.utils.LoggingSignalHandler) [2022-04-07 18:19:33,007] INFO [KafkaServer id=0] shutting down (kafka.server.KafkaServer) [2022-04-07 18:19:33,008] INFO [KafkaServer id=0] Starting controlled shutdown (kafka.server.KafkaServer) [2022-04-07 18:19:33,016] INFO [KafkaServer id=0] Controlled shutdown request returned successfully after 6ms (kafka.server.KafkaServer) [2022-04-07 18:19:33,227] INFO Broker and topic stats closed (kafka.server.BrokerTopicStats) [2022-04-07 18:19:33,227] INFO App info kafka.server for 0 unregistered (org.apache.kafka.common.utils.AppInfoParser) [2022-04-07 18:19:33,227] INFO [KafkaServer id=0] shut down completed (kafka.server.KafkaServer) // new added log [2022-04-07 18:19:33,227] INFO [KafkaServer id=0] the broker is shut down due to: signal SIGINT{code} was: Currently, when the broker hit exception, or SIGINT signal received, we will first log the exception (or the reason we started to shutdown) ex: _INFO Terminating process due to signal SIGINT_ And then go through the shutdown process, to close all the resources (i.e. threads, sockets, metrics...), and in the end, output: _[KafkaServer id=0] shut down completed (kafka.server.KafkaServer)_ Sometimes, during closing the resources, we will also hit some exceptions, or waiting for some other dependencies close, and output many more logs. In the end, when we tried to find out why the broker is shutdown, we have to go through many lines of logs from the tail, and identify the specific reason why the broker shutting down. We could improve it by adding the reason why the broker is shutting down at the end of log, something like this: {code:java} [2022-04-07 18:19:33,005] INFO Terminating process due to signal SIGINT (org.apache.kafka.common.utils.LoggingSignalHandler) [2022-04-07 18:19:33,007] INFO [KafkaServer id=0] shutting down (kafka.server.KafkaServer) [2022-04-07 18:19:33,008] INFO [KafkaServer id=0] Starting controlled shutdown (kafka.server.KafkaServer) [2022-04-07 18:19:33,016] INFO [KafkaServer id=0] Controlled shutdown request returned successfully after 6ms (kafka.server.KafkaServer) [2022-04-07 18:19:33,227] INFO Broker and topic stats closed (kafka.server.BrokerTopicStats) [2022-04-07 18:19:33,227] INFO App info kafka.server for 0 unregistered (org.apache.kafka.common.utils.AppInfoParser) [2022-04-07 18:19:33,227] INFO [KafkaServer id=0] shut down completed (kafka.server.KafkaServer) // new added log [2022-04-07 18:19:33,227] INFO [KafkaServer id=0] the broker is shut down due to: signal SIGINT{code} > Log broker shutdown reason at the end of log output > --- > > Key: KAFKA-13804 > URL: https://issues.apache.org/jira/browse/KAFKA-13804 > Project: Kafka > Issue Type: Improvement > Components: core >Reporter: Luke Chen >Assignee: Luke Chen >Priority: Major > > Currently, when the broker hit exception, or SIGINT signal received, we will > first log the exception (or the reason we started to shutdown) ex: > _INFO Terminating process due to signal SIGINT_ > > And then go through the shutdown process, to close all the resources (i.e. > threads, sockets, metrics...), and in the end, output: > _[KafkaServer id=0] shut down completed (kafka.server.KafkaServer)_ > > Sometimes, during closing the resources, we will also hit some exceptions, or > waiting for some other dependencies close, and output many more logs. When we > tried to find out why the broker is shutdown from the log, we have to go > through many lines of logs from the tail, and skip some unrelated exceptions, > and then identify the specific reaso
[jira] [Created] (KAFKA-13804) Log broker shutdown reason at the end of log output
Luke Chen created KAFKA-13804: - Summary: Log broker shutdown reason at the end of log output Key: KAFKA-13804 URL: https://issues.apache.org/jira/browse/KAFKA-13804 Project: Kafka Issue Type: Improvement Components: core Reporter: Luke Chen Assignee: Luke Chen Currently, when the broker hit exception, or SIGINT signal received, we will first log the exception (or the reason we started to shutdown) ex: _INFO Terminating process due to signal SIGINT_ And then go through the shutdown process, to close all the resources (i.e. threads, sockets, metrics...), and in the end, output: _[KafkaServer id=0] shut down completed (kafka.server.KafkaServer)_ Sometimes, during closing the resources, we will also hit some exceptions, or waiting for some other dependencies close, and output many more logs. In the end, when we tried to find out why the broker is shutdown, we have to go through many lines of logs from the tail, and identify the specific reason why the broker shutting down. We could improve it by adding the reason why the broker is shutting down at the end of log, something like this: {code:java} [2022-04-07 18:19:33,005] INFO Terminating process due to signal SIGINT (org.apache.kafka.common.utils.LoggingSignalHandler) [2022-04-07 18:19:33,007] INFO [KafkaServer id=0] shutting down (kafka.server.KafkaServer) [2022-04-07 18:19:33,008] INFO [KafkaServer id=0] Starting controlled shutdown (kafka.server.KafkaServer) [2022-04-07 18:19:33,016] INFO [KafkaServer id=0] Controlled shutdown request returned successfully after 6ms (kafka.server.KafkaServer) [2022-04-07 18:19:33,227] INFO Broker and topic stats closed (kafka.server.BrokerTopicStats) [2022-04-07 18:19:33,227] INFO App info kafka.server for 0 unregistered (org.apache.kafka.common.utils.AppInfoParser) [2022-04-07 18:19:33,227] INFO [KafkaServer id=0] shut down completed (kafka.server.KafkaServer) // new added log [2022-04-07 18:19:33,227] INFO [KafkaServer id=0] the broker is shut down due to: signal SIGINT{code} -- This message was sent by Atlassian Jira (v8.20.1#820001)
[GitHub] [kafka] EvansJahja opened a new pull request, #12011: MINOR: add print.topic property in console consumer
EvansJahja opened a new pull request, #12011: URL: https://github.com/apache/kafka/pull/12011 ``` Usage: ./kafka-console-consumer.sh --bootstrap-server kafka:9092 --include '.*' --property print.topic=true Example output: Topic:hello.world{"msg": "abc"} ``` Purpose: Until now, there is no way to print all messages from all topics while maintaining the topic information. -- 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.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org
[GitHub] [kafka] showuon merged pull request #12007: MINOR: Fix method javadoc and typo in comments
showuon merged PR #12007: URL: https://github.com/apache/kafka/pull/12007 -- 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.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org
[GitHub] [kafka] showuon commented on pull request #12007: MINOR: Fix method annotation
showuon commented on PR #12007: URL: https://github.com/apache/kafka/pull/12007#issuecomment-1091493302 Failed tests are unrelated: ``` Build / ARM / kafka.network.ConnectionQuotasTest.testListenerConnectionRateLimitWhenActualRateAboveLimit() ``` -- 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.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org
[jira] [Commented] (KAFKA-13793) Add validators for serialization and deserialization related configuration
[ https://issues.apache.org/jira/browse/KAFKA-13793?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17518732#comment-17518732 ] RivenSun commented on KAFKA-13793: -- Hi [~showuon] , please help review PR-12010. Thanks. > Add validators for serialization and deserialization related configuration > -- > > Key: KAFKA-13793 > URL: https://issues.apache.org/jira/browse/KAFKA-13793 > Project: Kafka > Issue Type: Improvement > Components: clients, config >Reporter: RivenSun >Assignee: RivenSun >Priority: Major > > These configurations of producer and consumer have the same problem. > {code:java} > key.serializer, value.serializer, key.deserializer, value.deserializer{code} > > Take the `key.serializer` configuration as an example: > {code:java} > Map props = new HashMap<>(); > props.put("key.serializer", null);{code} > It is expected that this abnormal configuration can be verified during the > startup process of kafkaProducer, but the actual startup result: > {code:java} > Exception in thread "main" org.apache.kafka.common.KafkaException: Failed to > construct kafka producer > at > org.apache.kafka.clients.producer.KafkaProducer.(KafkaProducer.java:440) > at > org.apache.kafka.clients.producer.KafkaProducer.(KafkaProducer.java:291) > at > org.apache.kafka.clients.producer.KafkaProducer.(KafkaProducer.java:274) > at > us.zoom.mq.server.adapter.kafka.ProducerTest.main(ProducerTest.java:139) > Caused by: java.lang.NullPointerException > at > org.apache.kafka.clients.producer.KafkaProducer.(KafkaProducer.java:368) > ... 3 more {code} > There was a line of code that threw a null pointer, causing KafkaProducer > initialization to fail. > I think we should be able to find this bad configuration during the > validation of all the configuration i.e. execute the > *ConfigDef.parseValue(ConfigKey key, Object value, boolean isSet) method* and > throw a *ConfigException* instead of NullPointerException. > Solution: > Add *NonNullValidator* to these configurations. > For example, when ProducerConfig defines `key.serializer` configuration, add > Validator: > {code:java} > .define(KEY_SERIALIZER_CLASS_CONFIG, > Type.CLASS, > ConfigDef.NO_DEFAULT_VALUE, > new ConfigDef.NonNullValidator(), > Importance.HIGH, > KEY_SERIALIZER_CLASS_DOC) {code} > -- This message was sent by Atlassian Jira (v8.20.1#820001)
[GitHub] [kafka] bozhao12 closed pull request #12009: MINOR: Fix method annotation
bozhao12 closed pull request #12009: MINOR: Fix method annotation URL: https://github.com/apache/kafka/pull/12009 -- 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.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org
[GitHub] [kafka] RivenSun2 commented on pull request #12010: KAFKA-13793: Add validators for configs that lack validators
RivenSun2 commented on PR #12010: URL: https://github.com/apache/kafka/pull/12010#issuecomment-1091397361 Hi @showuon could you help to review the PR? Thanks. -- 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.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org
[GitHub] [kafka] RivenSun2 opened a new pull request, #12010: KAFKA-13793: Add validators for configs that lack validators
RivenSun2 opened a new pull request, #12010: URL: https://github.com/apache/kafka/pull/12010 Add validators for configs that lack validators -- 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.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org
[GitHub] [kafka] bozhao12 commented on pull request #12009: MINOR: Fix method annotation
bozhao12 commented on PR #12009: URL: https://github.com/apache/kafka/pull/12009#issuecomment-1091395431 @showuon Sorry, I deleted the remote branch by mistake, I submitted a new Pull Request. Thanks -- 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.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org
[GitHub] [kafka] bozhao12 opened a new pull request, #12009: MINOR: fix typo and method annotation
bozhao12 opened a new pull request, #12009: URL: https://github.com/apache/kafka/pull/12009 ### Committer Checklist (excluded from commit message) - [ ] Verify design and implementation - [ ] Verify test coverage and CI build status - [ ] Verify documentation (including upgrade notes) -- 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.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org
[GitHub] [kafka] bozhao12 closed pull request #12007: MINOR: Fix method annotation
bozhao12 closed pull request #12007: MINOR: Fix method annotation URL: https://github.com/apache/kafka/pull/12007 -- 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.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org
[GitHub] [kafka] ddrid commented on a diff in pull request #12006: KAFKA-13794: Follow up to fix comparator
ddrid commented on code in PR #12006: URL: https://github.com/apache/kafka/pull/12006#discussion_r844881404 ## clients/src/main/java/org/apache/kafka/clients/producer/internals/TransactionManager.java: ## @@ -187,7 +187,7 @@ private void startSequencesAtBeginning(TopicPartition topicPartition, ProducerId private static final Comparator PRODUCER_BATCH_COMPARATOR = (b1, b2) -> { if (b1.baseSequence() < b2.baseSequence()) return -1; else if (b1.baseSequence() > b2.baseSequence()) return 1; -else return b1.equals(b2) ? 0 : 1; +else return b1.equals(b2) ? 0 : Integer.compare(b1.hashCode(), b2.hashCode()); Review Comment: Hi @showuon, thanks for you comment. I found some trouble when adding `equals` method in `ProducerBatch`. The only two final fields I can use is `createdMs` and `topicPartition`, I don't think it make sense if I use these two fields since they usually have duplicates. Do you have some suggestions? Thanks -- 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.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org
[GitHub] [kafka] showuon commented on pull request #11971: KAFKA-13783; Remove reason prefixing in JoinGroupRequest and LeaveGroupRequest
showuon commented on PR #11971: URL: https://github.com/apache/kafka/pull/11971#issuecomment-1091344371 > We have found the root cause of the original issue. It was a bug in our benchmark setup so it had nothing to do with this change in the end. We can still keep this change as it looks better to me like this. Thanks for the update. Great to hear that! -- 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.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org
[jira] [Resolved] (KAFKA-13801) Kafka server does not respect MetricsReporter interface contract for dynamically configured reporters
[ https://issues.apache.org/jira/browse/KAFKA-13801?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] David Jacot resolved KAFKA-13801. - Fix Version/s: 3.3.0 Reviewer: David Jacot Assignee: Xavier Léauté Resolution: Fixed > Kafka server does not respect MetricsReporter interface contract for > dynamically configured reporters > - > > Key: KAFKA-13801 > URL: https://issues.apache.org/jira/browse/KAFKA-13801 > Project: Kafka > Issue Type: Bug > Components: metrics >Reporter: Xavier Léauté >Assignee: Xavier Léauté >Priority: Minor > Fix For: 3.3.0 > > > MetricsReporter.contextChange contract states the method should always > be called first before MetricsReporter.init is called. This is done > correctly for reporters enabled by default (e.g. JmxReporter) but not > for metrics reporters configured dynamically -- This message was sent by Atlassian Jira (v8.20.1#820001)
[GitHub] [kafka] dajac merged pull request #11998: KAFKA-13801: Kafka server does not respect MetricsReporter contract for dynamically configured reporters
dajac merged PR #11998: URL: https://github.com/apache/kafka/pull/11998 -- 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.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org
[jira] [Commented] (KAFKA-13793) Add validators for serialization and deserialization related configuration
[ https://issues.apache.org/jira/browse/KAFKA-13793?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17518660#comment-17518660 ] RivenSun commented on KAFKA-13793: -- Kafka itself supports five sasl mechanisms, GSSAPI (Kerberos), PLAIN, SCRAM-SHA-256, SCRAM-SHA-512, OAUTHBEARER But Kafka's sasl module is highly flexible and customizable. The value of `sasl.mechanism` is not limited to the above five mechanisms {code:java} sasl.mechanism sasl.enabled.mechanisms sasl.mechanism.controller.protocol sasl.mechanism.inter.broker.protocol{code} Therefore, for the validators of these configuration items, the validation element cannot be null or an empty string. > Add validators for serialization and deserialization related configuration > -- > > Key: KAFKA-13793 > URL: https://issues.apache.org/jira/browse/KAFKA-13793 > Project: Kafka > Issue Type: Improvement > Components: clients, config >Reporter: RivenSun >Assignee: RivenSun >Priority: Major > > These configurations of producer and consumer have the same problem. > {code:java} > key.serializer, value.serializer, key.deserializer, value.deserializer{code} > > Take the `key.serializer` configuration as an example: > {code:java} > Map props = new HashMap<>(); > props.put("key.serializer", null);{code} > It is expected that this abnormal configuration can be verified during the > startup process of kafkaProducer, but the actual startup result: > {code:java} > Exception in thread "main" org.apache.kafka.common.KafkaException: Failed to > construct kafka producer > at > org.apache.kafka.clients.producer.KafkaProducer.(KafkaProducer.java:440) > at > org.apache.kafka.clients.producer.KafkaProducer.(KafkaProducer.java:291) > at > org.apache.kafka.clients.producer.KafkaProducer.(KafkaProducer.java:274) > at > us.zoom.mq.server.adapter.kafka.ProducerTest.main(ProducerTest.java:139) > Caused by: java.lang.NullPointerException > at > org.apache.kafka.clients.producer.KafkaProducer.(KafkaProducer.java:368) > ... 3 more {code} > There was a line of code that threw a null pointer, causing KafkaProducer > initialization to fail. > I think we should be able to find this bad configuration during the > validation of all the configuration i.e. execute the > *ConfigDef.parseValue(ConfigKey key, Object value, boolean isSet) method* and > throw a *ConfigException* instead of NullPointerException. > Solution: > Add *NonNullValidator* to these configurations. > For example, when ProducerConfig defines `key.serializer` configuration, add > Validator: > {code:java} > .define(KEY_SERIALIZER_CLASS_CONFIG, > Type.CLASS, > ConfigDef.NO_DEFAULT_VALUE, > new ConfigDef.NonNullValidator(), > Importance.HIGH, > KEY_SERIALIZER_CLASS_DOC) {code} > -- This message was sent by Atlassian Jira (v8.20.1#820001)
[GitHub] [kafka] dajac commented on pull request #11971: KAFKA-13783; Remove reason prefixing in JoinGroupRequest and LeaveGroupRequest
dajac commented on PR #11971: URL: https://github.com/apache/kafka/pull/11971#issuecomment-1091191284 We have found the root cause of the original issue. It was a bug in our benchmark setup so it had nothing to do with this change in the end. We can still keep this change as it looks better to me like this. -- 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.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org