vvcephei commented on a change in pull request #10507:
URL: https://github.com/apache/kafka/pull/10507#discussion_r636995037



##########
File path: 
streams/src/main/java/org/apache/kafka/streams/errors/DeserializationExceptionHandler.java
##########
@@ -31,14 +32,14 @@
      * Inspect a record and the exception received.
      * <p>
      * Note, that the passed in {@link ProcessorContext} only allows to access 
metadata like the task ID.
-     * However, it cannot be used to emit records via {@link 
ProcessorContext#forward(Object, Object)};
+     * However, it cannot be used to emit records via {@link 
ProcessorContext#forward(Record)};
      * calling {@code forward()} (and some other methods) would result in a 
runtime exception.
      *
      * @param context processor context
      * @param record record that failed deserialization
      * @param exception the actual exception
      */
-    DeserializationHandlerResponse handle(final ProcessorContext context,
+    DeserializationHandlerResponse handle(final ProcessorContext<?, ?> context,

Review comment:
       Oh, man. I overlooked this in the KIP, and we can't just change this 
in-place, as it will break any subclasses.
   
   What we need to do is deprecate this method and introduce a new one with a 
default implementation that calls back here. We can update the KIP with this 
change, since it's a simple oversight and follows established patterns for 
migrating interfaces.

##########
File path: streams/src/main/java/org/apache/kafka/streams/kstream/KTable.java
##########
@@ -16,6 +16,7 @@
  */
 package org.apache.kafka.streams.kstream;
 
+import java.util.function.Function;
 import org.apache.kafka.common.serialization.Serde;

Review comment:
       Also, the same question here: do we need any changes to this interface?

##########
File path: 
streams/src/main/java/org/apache/kafka/streams/kstream/ValueJoiner.java
##########
@@ -23,9 +23,9 @@
  * record-pair of a {@link KStream}-{@link KStream}, {@link KStream}-{@link 
KTable}, or {@link KTable}-{@link KTable}
  * join.
  *
- * @param <V1> first value type
- * @param <V2> second value type
- * @param <VR> joined value type

Review comment:
       Also here: it doesn't seem strictly necessary to rename the generic 
parameters as part of this PR.

##########
File path: streams/src/main/java/org/apache/kafka/streams/kstream/KStream.java
##########
@@ -26,29 +26,29 @@
 import org.apache.kafka.streams.Topology;
 import org.apache.kafka.streams.processor.ConnectedStoreProvider;
 import org.apache.kafka.streams.processor.Processor;
-import org.apache.kafka.streams.processor.ProcessorContext;
-import org.apache.kafka.streams.processor.ProcessorSupplier;
 import org.apache.kafka.streams.processor.StreamPartitioner;
 import org.apache.kafka.streams.processor.TopicNameExtractor;
+import org.apache.kafka.streams.processor.api.ProcessorContext;
+import org.apache.kafka.streams.processor.api.ProcessorSupplier;
 import org.apache.kafka.streams.state.KeyValueStore;

Review comment:
       Hey @jeqo , based on my understanding of the KIP, nothing in this 
interface should have changed. Was this changeset intentional?




-- 
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


Reply via email to