mjsax commented on a change in pull request #10573: URL: https://github.com/apache/kafka/pull/10573#discussion_r619504008
########## File path: clients/src/main/java/org/apache/kafka/clients/producer/KafkaProducer.java ########## @@ -642,7 +647,10 @@ public void beginTransaction() throws ProducerFencedException { * to the partition leader. See the exception for more details * @throws KafkaException if the producer has encountered a previous fatal or abortable error, or for any * other unexpected error + * + * @deprecated Since 3.0.0, will be removed in 4.0. Use {@link #sendOffsetsToTransaction(Map, ConsumerGroupMetadata)} instead. Review comment: I don't think we will be able to remove it in 4.0 and I would assume that 4.0 is too early, and we usually keep API around for at least 1 year after deprecation. Maybe best to just remove this part? ########## File path: docs/streams/developer-guide/config-streams.html ########## @@ -293,8 +293,9 @@ <h4><a class="toc-backref" href="#id5">bootstrap.servers</a><a class="headerlink </tr> <tr class="row-even"><td>processing.guarantee</td> <td>Medium</td> - <td colspan="2">The processing mode. Can be either <code class="docutils literal"><span class="pre">"at_least_once"</span></code> (default), - <code class="docutils literal"><span class="pre">"exactly_once"</span></code> (for EOS version 1), or <code class="docutils literal"><span class="pre">"exactly_once_beta"</span></code> (for EOS version 2)</td>. + <td colspan="2">The processing mode. Can be either <code class="docutils literal"><span class="pre">"at_least_once"</span></code> (default) + or <code class="docutils literal"><span class="pre">"exactly_once_v2"</span></code> (for EOS version 2, requires broker version 2.5+). Deprecated config options are + <code class="docutils literal"><span class="pre">"exactly_once"</span></code> (for EOS version 1) and <code class="docutils literal"><span class="pre">"exactly_once_beta"</span></code> (for EOS beta, requires broker version 2.5+)</td>. Review comment: nit `for EOS beta` -> `for EOS version 2` (Might be really good to get rid of the term "beta" whenever we can -- and also make it more clear that we just renamed this config.) ########## File path: docs/streams/upgrade-guide.html ########## @@ -93,6 +95,12 @@ <h1>Upgrade Guide and API Changes</h1> </p> <h3><a id="streams_api_changes_300" href="#streams_api_changes_300">Streams API changes in 3.0.0</a></h3> + <p> + The <code>StreamsConfig.EXACTLY_ONCE</code> and <code>StreamsConfig.EXACTLY_ONCE_BETA</code> configs have been deprecated, and a new <code>StreamsConfig.EXACTLY_ONCE_V2</code> config has been + introduced. This is the same feature as eos-beta, but renamed to highlight its production-readiness. Users of exactly-once semantics should plan to migrate to the eos-v2 config and prepare for the removal of the deprecated configs in 4.0 or after at least a year + from the release of 3.0, whichever comes last. Note that eos-v2 requires broker version 2.5 or higher, like eos-beta, so users should begin to upgrade their kafka cluster if necessary. See Review comment: Personally, I would not talk about when we plan to remove stuff, because the plan might change. ########## File path: streams/src/main/java/org/apache/kafka/streams/StreamsConfig.java ########## @@ -291,23 +293,35 @@ * <p> * Enabling exactly-once processing semantics requires broker version 0.11.0 or higher. * If you enable this feature Kafka Streams will use more resources (like broker connections) - * compared to the {@link #AT_LEAST_ONCE} case. + * compared to {@link #AT_LEAST_ONCE "at_least_once"} and {@link #EXACTLY_ONCE_V2 "exactly_once_v2"}. * - * @see #EXACTLY_ONCE_BETA + * @deprecated Since 3.0.0, will be removed in 4.0. Use {@link #EXACTLY_ONCE_V2 "exactly_once_v2"} instead. */ @SuppressWarnings("WeakerAccess") + @Deprecated public static final String EXACTLY_ONCE = "exactly_once"; /** * Config value for parameter {@link #PROCESSING_GUARANTEE_CONFIG "processing.guarantee"} for exactly-once processing guarantees. * <p> * Enabling exactly-once (beta) requires broker version 2.5 or higher. - * If you enable this feature Kafka Streams will use less resources (like broker connections) - * compare to the {@link #EXACTLY_ONCE} case. + * If you enable this feature Kafka Streams will use fewer resources (like broker connections) + * compared to the {@link #EXACTLY_ONCE} case. Review comment: `{@link #EXACTLY_ONCE} (deprecated) case.` ########## File path: docs/streams/developer-guide/config-streams.html ########## @@ -667,12 +668,14 @@ <h4><a class="toc-backref" href="#id30">probing.rebalance.interval.ms</a><a clas <span id="streams-developer-guide-processing-guarantee"></span><h4><a class="toc-backref" href="#id25">processing.guarantee</a><a class="headerlink" href="#processing-guarantee" title="Permalink to this headline"></a></h4> <blockquote> <div>The processing guarantee that should be used. - Possible values are <code class="docutils literal"><span class="pre">"at_least_once"</span></code> (default), - <code class="docutils literal"><span class="pre">"exactly_once"</span></code> (for EOS version 1), - and <code class="docutils literal"><span class="pre">"exactly_once_beta"</span></code> (for EOS version 2). - Using <code class="docutils literal"><span class="pre">"exactly_once"</span></code> requires broker - version 0.11.0 or newer, while using <code class="docutils literal"><span class="pre">"exactly_once_beta"</span></code> - requires broker version 2.5 or newer. + Possible values are <code class="docutils literal"><span class="pre">"at_least_once"</span></code> (default) + and <code class="docutils literal"><span class="pre">"exactly_once_v2"</span></code> (for EOS version 2). + Deprecated config options are <code class="docutils literal"><span class="pre">"exactly_once"</span></code> (for EOS alpha), Review comment: Fair enough. ########## File path: docs/streams/developer-guide/config-streams.html ########## @@ -667,12 +668,14 @@ <h4><a class="toc-backref" href="#id30">probing.rebalance.interval.ms</a><a clas <span id="streams-developer-guide-processing-guarantee"></span><h4><a class="toc-backref" href="#id25">processing.guarantee</a><a class="headerlink" href="#processing-guarantee" title="Permalink to this headline"></a></h4> <blockquote> <div>The processing guarantee that should be used. - Possible values are <code class="docutils literal"><span class="pre">"at_least_once"</span></code> (default), - <code class="docutils literal"><span class="pre">"exactly_once"</span></code> (for EOS version 1), - and <code class="docutils literal"><span class="pre">"exactly_once_beta"</span></code> (for EOS version 2). - Using <code class="docutils literal"><span class="pre">"exactly_once"</span></code> requires broker - version 0.11.0 or newer, while using <code class="docutils literal"><span class="pre">"exactly_once_beta"</span></code> - requires broker version 2.5 or newer. + Possible values are <code class="docutils literal"><span class="pre">"at_least_once"</span></code> (default) + and <code class="docutils literal"><span class="pre">"exactly_once_v2"</span></code> (for EOS version 2). + Deprecated config options are <code class="docutils literal"><span class="pre">"exactly_once"</span></code> (for EOS alpha), + and <code class="docutils literal"><span class="pre">"exactly_once_beta"</span></code> (for EOS beta). Review comment: as above: `beta` -> `version 2` ########## File path: clients/src/main/java/org/apache/kafka/clients/producer/KafkaProducer.java ########## @@ -657,8 +665,10 @@ public void sendOffsetsToTransaction(Map<TopicPartition, OffsetAndMetadata> offs * This method should be used when you need to batch consumed and produced messages * together, typically in a consume-transform-produce pattern. Thus, the specified * {@code groupMetadata} should be extracted from the used {@link KafkaConsumer consumer} via - * {@link KafkaConsumer#groupMetadata()} to leverage consumer group metadata for stronger fencing than - * {@link #sendOffsetsToTransaction(Map, String)} which only sends with consumer group id. + * {@link KafkaConsumer#groupMetadata()} to leverage consumer group metadata. This will provide + * stronger fencing than just supplying the consumerGroupId and passing in {@code new ConsumerGroupMetadata(consumerGroupId)}, Review comment: nit `{@code consumerGroupId}` ########## File path: docs/streams/upgrade-guide.html ########## @@ -53,17 +53,19 @@ <h1>Upgrade Guide and API Changes</h1> </ul> <p> - Starting in Kafka Streams 2.6.x, a new processing mode is available, named EOS version 2, which is configurable by setting - <code>processing.guarantee</code> to <code>"exactly_once_beta"</code>. - <b>NOTE:</b> The <code>"exactly_once_beta"</code> processing mode is ready for production (<i>i.e.</i>, it's not "beta" software). + Starting in Kafka Streams 2.6.x, a new processing mode is available, named EOS version 2. This can be configured + by setting <code>StreamsConfig.PROCESSING_GUARANTEE</code> to <code>StreamsConfig.EXACTLY_ONCE_V2</code> for + application versions 3.0+, or setting it to <code>StreamsConfig.EXACTLY_ONCE_BETA</code> for versions between 2.6 and 2.8. To use this new feature, your brokers must be on version 2.5.x or newer. - A switch from <code>"exactly_once"</code> to <code>"exactly_once_beta"</code> (or the other way around) is - only possible if the application is on version 2.6.x. - If you want to upgrade your application from an older version and enable this feature, - you first need to upgrade your application to version 2.6.x, staying on <code>"exactly_once"</code>, - and then do second round of rolling bounces to switch to <code>"exactly_once_beta"</code>. - For a downgrade, do the reverse: first switch the config from <code>"exactly_once_beta"</code> to - <code>"exactly_once"</code> to disable the feature in your 2.6.x application. + If you want to upgrade your EOS application from an older version and enable this feature in version 3.0+, + you first need to upgrade your application to version 3.0.x, staying on <code>StreamsConfig.EXACTLY_ONCE</code>, + and then do second round of rolling bounces to switch to <code>StreamsConfig.EXACTLY_ONCE_V2</code>. If you + are upgrading an EOS application from an older (pre-2.6) version to a version between 2.6 and 2.8, follow these + same steps but with the config <code>StreamsConfig.EXACTLY_ONCE_BETA</code> instead. No special steps are required + to upgrade an application using <code>StreamsConfig.EXACTLY_ONCE_BETA</code> from version 2.6+ to 3.0 or higher: you can + just replace the config during the rolling upgrade. Review comment: `replace the config` -> `change the config from <code>StreamsConfig.EXACTLY_ONCE_BETA</code> to <code>StreamsConfig.EXACTLY_ONCE_V2</code> during the rolling upgrade.` (It might sound redundant, but in upgrade notes one cannot be too precise.) ########## File path: docs/streams/upgrade-guide.html ########## @@ -53,17 +53,19 @@ <h1>Upgrade Guide and API Changes</h1> </ul> <p> - Starting in Kafka Streams 2.6.x, a new processing mode is available, named EOS version 2, which is configurable by setting - <code>processing.guarantee</code> to <code>"exactly_once_beta"</code>. - <b>NOTE:</b> The <code>"exactly_once_beta"</code> processing mode is ready for production (<i>i.e.</i>, it's not "beta" software). + Starting in Kafka Streams 2.6.x, a new processing mode is available, named EOS version 2. This can be configured + by setting <code>StreamsConfig.PROCESSING_GUARANTEE</code> to <code>StreamsConfig.EXACTLY_ONCE_V2</code> for + application versions 3.0+, or setting it to <code>StreamsConfig.EXACTLY_ONCE_BETA</code> for versions between 2.6 and 2.8. To use this new feature, your brokers must be on version 2.5.x or newer. - A switch from <code>"exactly_once"</code> to <code>"exactly_once_beta"</code> (or the other way around) is - only possible if the application is on version 2.6.x. - If you want to upgrade your application from an older version and enable this feature, - you first need to upgrade your application to version 2.6.x, staying on <code>"exactly_once"</code>, - and then do second round of rolling bounces to switch to <code>"exactly_once_beta"</code>. - For a downgrade, do the reverse: first switch the config from <code>"exactly_once_beta"</code> to - <code>"exactly_once"</code> to disable the feature in your 2.6.x application. + If you want to upgrade your EOS application from an older version and enable this feature in version 3.0+, + you first need to upgrade your application to version 3.0.x, staying on <code>StreamsConfig.EXACTLY_ONCE</code>, Review comment: Why do we switch from `"exaclty_once"` to `StreamsConfig.EXACTLY_ONCE`? If users have a config properties / text file, you would use the string... ########## File path: docs/streams/upgrade-guide.html ########## @@ -93,6 +95,12 @@ <h1>Upgrade Guide and API Changes</h1> </p> <h3><a id="streams_api_changes_300" href="#streams_api_changes_300">Streams API changes in 3.0.0</a></h3> + <p> + The <code>StreamsConfig.EXACTLY_ONCE</code> and <code>StreamsConfig.EXACTLY_ONCE_BETA</code> configs have been deprecated, and a new <code>StreamsConfig.EXACTLY_ONCE_V2</code> config has been + introduced. This is the same feature as eos-beta, but renamed to highlight its production-readiness. Users of exactly-once semantics should plan to migrate to the eos-v2 config and prepare for the removal of the deprecated configs in 4.0 or after at least a year Review comment: > to highlight its production-readiness Sound like it becomes production ready in 3.0 release? ########## File path: streams/src/main/java/org/apache/kafka/streams/StreamsConfig.java ########## @@ -291,23 +293,35 @@ * <p> * Enabling exactly-once processing semantics requires broker version 0.11.0 or higher. * If you enable this feature Kafka Streams will use more resources (like broker connections) - * compared to the {@link #AT_LEAST_ONCE} case. + * compared to {@link #AT_LEAST_ONCE "at_least_once"} and {@link #EXACTLY_ONCE_V2 "exactly_once_v2"}. * - * @see #EXACTLY_ONCE_BETA + * @deprecated Since 3.0.0, will be removed in 4.0. Use {@link #EXACTLY_ONCE_V2 "exactly_once_v2"} instead. Review comment: remove `will be removed in 4.0` (similar elsewhere) ########## File path: docs/streams/upgrade-guide.html ########## @@ -93,6 +95,12 @@ <h1>Upgrade Guide and API Changes</h1> </p> <h3><a id="streams_api_changes_300" href="#streams_api_changes_300">Streams API changes in 3.0.0</a></h3> + <p> + The <code>StreamsConfig.EXACTLY_ONCE</code> and <code>StreamsConfig.EXACTLY_ONCE_BETA</code> configs have been deprecated, and a new <code>StreamsConfig.EXACTLY_ONCE_V2</code> config has been Review comment: As above: should we use `"exaclty_once"` instead of `StreamConfig.EXACTLY_ONCE` ? ########## File path: streams/src/main/java/org/apache/kafka/streams/processor/internals/StreamThread.java ########## @@ -603,7 +606,7 @@ boolean runLoop() { log.error("Shutting down because the Kafka cluster seems to be on a too old version. " + "Setting {}=\"{}\" requires broker version 2.5 or higher.", StreamsConfig.PROCESSING_GUARANTEE_CONFIG, - EXACTLY_ONCE_BETA); + StreamsConfig.EXACTLY_ONCE_V2); Review comment: We don't know if the user set `_beta` or `_v2` -- should we try to get the value from `StreamsConfig` to provide an error message that matches whatever the user specified? ########## File path: streams/src/test/java/org/apache/kafka/streams/integration/ResetPartitionTimeIntegrationTest.java ########## @@ -91,6 +91,7 @@ public static void closeCluster() { private static final int DEFAULT_TIMEOUT = 100; private static long lastRecordedTimestamp = -2L; + @SuppressWarnings("deprecation") Review comment: nit (below) replace `_beta` with `_v2` (even if it does not avoid the need to suppress warnings... ########## File path: streams/src/test/java/org/apache/kafka/streams/processor/internals/RecordCollectorTest.java ########## @@ -111,6 +113,7 @@ private final StringSerializer stringSerializer = new StringSerializer(); private final ByteArraySerializer byteArraySerializer = new ByteArraySerializer(); + private final UUID processId = UUID.randomUUID(); Review comment: Why do we need to introduce this one? ########## File path: streams/src/main/java/org/apache/kafka/streams/StreamsConfig.java ########## @@ -122,7 +122,9 @@ * <li>{@link ConsumerConfig#PARTITION_ASSIGNMENT_STRATEGY_CONFIG "partition.assignment.strategy"} (<code>StreamsPartitionAssignor</code>) - Streams client will always use its own partition assignor</li> * </ul> * - * If {@link #PROCESSING_GUARANTEE_CONFIG "processing.guarantee"} is set to {@link #EXACTLY_ONCE "exactly_once"}, Kafka Streams does not allow users to overwrite the following properties (Streams setting shown in parentheses): + * If {@link #PROCESSING_GUARANTEE_CONFIG "processing.guarantee"} is set to {@link #EXACTLY_ONCE_V2 "exactly_once_v2"}, + * {@link #EXACTLY_ONCE "exactly_once"}, or {@link #EXACTLY_ONCE_BETA "exactly_once_beta"}, Kafka Streams does not Review comment: `{@link #EXACTLY_ONCE "exactly_once"} (deprecated), or {@link #EXACTLY_ONCE_BETA "exactly_once_beta"} (deprecated)` ########## File path: streams/src/main/java/org/apache/kafka/streams/processor/internals/ActiveTaskCreator.java ########## @@ -115,7 +115,7 @@ public void reInitializeThreadProducer() { StreamsProducer streamsProducerForTask(final TaskId taskId) { if (processingMode != EXACTLY_ONCE_ALPHA) { - throw new IllegalStateException("Producer per thread is used."); + throw new IllegalStateException("Expected EXACTLY_ONCE to be enabled, but the processing mode was " + processingMode); Review comment: nit: `EXACTLY_ONCE` -> `eos-v1` (to align to `eos-v2` below) ########## File path: streams/src/test/java/org/apache/kafka/streams/processor/internals/RecordCollectorTest.java ########## @@ -126,10 +129,10 @@ public void setup() { clientSupplier.setCluster(cluster); streamsProducer = new StreamsProducer( config, - "threadId", + processId + "-StreamThread-1", Review comment: Just hard code `"-StreamThread-1"` -- why do we need to prefix it? We used a hard-coded value before. -- 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. For queries about this service, please contact Infrastructure at: us...@infra.apache.org