-
Notifications
You must be signed in to change notification settings - Fork 13.5k
New issue
Have a question about this project? Sign up for a free GitHub account to open an issue and contact its maintainers and the community.
By clicking “Sign up for GitHub”, you agree to our terms of service and privacy statement. We’ll occasionally send you account related emails.
Already on GitHub? Sign in to your account
[FLINK-37521][runtime] Introduce KeyedCoProcessOperator with async state API #26328
base: master
Are you sure you want to change the base?
Conversation
import org.apache.flink.util.function.ThrowingConsumer; | ||
|
||
/** | ||
* A function that processes elements of two keyed streams and produces a single output one. |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
nit: one. -> stream.
* <p>The function will be called for every element in the input streams and can produce zero or | ||
* more output elements. Contrary to the {@link CoFlatMapFunction}, this function can also query the | ||
* time (both event and processing) and set timers, through the provided {@link Context}. When | ||
* reacting to the firing of set timers the function can emit yet more elements. |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
nit. in the previous sentence I assume that this function can set timers. In this sentence set timers is not so clear. Maybe "When reacting to the firing of timers," or "When reacting to the firing of timers that have been set,"
* time (both event and processing) and set timers, through the provided {@link Context}. When | ||
* reacting to the firing of set timers the function can emit yet more elements. | ||
* | ||
* <p>An example use-case for connected streams would be the application of a set of rules that |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
nits:
use-case -> use case
would be -> is
* change over time ({@code stream A}) to the elements contained in another stream (stream {@code | ||
* B}). The rules contained in {@code stream A} can be stored in the state and wait for new elements | ||
* to arrive on {@code stream B}. Upon reception of a new element on {@code stream B}, the function | ||
* can now apply the previously stored rules to the element and directly emit a result, and/or |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
nits:
remove the word now
remove the word directly
* @throws Exception The function may throw exceptions which cause the streaming program to fail | ||
* and go into recovery. | ||
*/ | ||
public void processElement1(IN1 value, Context ctx, Collector<OUT> out) throws Exception { |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
nit: processElement1 -> processInputStream1Element or the like
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
We can't rename it, because processElement1
is inherited from KeyedCoProcessFunction
. I added override
here.
* and go into recovery. | ||
*/ | ||
public void processElement1(IN1 value, Context ctx, Collector<OUT> out) throws Exception { | ||
throw new IllegalAccessException("This method is replaced by declareProcess1."); |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
what does this exception message mean?
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
This method is inherited from KeyedCoProcessFunction, but we want to use declareProcess1
to replace it.
This method should not be called as expected, so an exception thrown to prevent misuse.
* <p>This function can output zero or more elements using the {@link Collector} parameter and | ||
* also update internal state or set timers using the {@link Context} parameter. | ||
* | ||
* @param value The stream element |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
why is this parameter called value? Could it be something more specific like streamElement
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
Same as the question above, this method is inherited from KeyedCoProcessFunction
, the naming is consistent with KeyedCoProcessFunction
. I prefer keeping it.
* also update internal state or set timers using the {@link Context} parameter. | ||
* | ||
* @param value The stream element | ||
* @param ctx A {@link Context} that allows querying the timestamp of the element, querying the |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
nit: maybe a list would be easier to read in the javadoc. A {@link Context} that allows querying of
-
...
When you say querying the time
is this event or processing?
* timers and querying the time. The context is only valid during the invocation of this | ||
* method, do not store it. | ||
* @param out The collector to emit resulting elements to | ||
* @throws Exception The function may throw exceptions which cause the streaming program to fail |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
can it throw a retriable exception?
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
What is a retriable exception
?
...org/apache/flink/runtime/asyncprocessing/functions/DeclaringAsyncKeyedCoProcessFunction.java
Outdated
Show resolved
Hide resolved
* @throws Exception The function may throw exceptions which cause the streaming program to fail | ||
* and go into recovery. | ||
*/ | ||
public void processElement1(IN1 value, Context ctx, Collector<OUT> out) throws Exception { |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
we should label all the method as @public or @experimental or the like.
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
I don't think this is necessary because it is inherited from KeyedCoProcessFunction
. I add override
here.
throws DeclarationException; | ||
|
||
/** | ||
* Declare a procedure which is called when a timer set using {@link TimerService} fires. |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
the words say it is declaring a procedure but the output parameter says it is a processor. I would expect these words to be the same. The method name should align with what we are declaring. maybe declareStream2Processor
* of this method, do not store it. | ||
* @param out The processor for processing timestamps. | ||
*/ | ||
public ThrowingConsumer<Long, Exception> declareOnTimer( |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
nit: declareOnTimer -> declareOnTimerProcessor
private static final long serialVersionUID = 1L; | ||
|
||
// Shared timestamp variable for collector, context and onTimerContext. | ||
private transient DeclaredVariable<Long> sharedTimestamp; |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
what do we mean by the sharedTimestamp? Is this event time?
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
It is used to store one Timestamp. It can be event time or processing time.
collector = new TimestampedCollectorWithDeclaredVariable<>(output, sharedTimestamp); | ||
|
||
InternalTimerService<VoidNamespace> internalTimerService = | ||
getInternalTimerService("user-timers", VoidNamespaceSerializer.INSTANCE, this); |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
nit: maybe change to a constant and defined in the same place as other named InternalTimerServices. I am not sure what does user mean this context.
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
The naming "user-timers" in AsyncKeyedCoProcessOperator
is consistent with KeyedCoProcessOperator#internalTimerService
. It represents a timer-service
created by operator.
if (userFunction instanceof DeclaringAsyncKeyedCoProcessFunction) { | ||
DeclaringAsyncKeyedCoProcessFunction declaringFunction = | ||
(DeclaringAsyncKeyedCoProcessFunction) userFunction; | ||
declaringFunction.declareVariables(declarationContext); |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
I am curious, is there a description of what these declarations are and why we are declaring things for async?
I assume we are describing the variables, functions and timers that will be realised when they run asynchronously.
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
You can find the description about declarations at https://cwiki.apache.org/confluence/display/FLINK/FLIP-455%3A+Declare+async+state+processing+and+checkpoint+the+in-flight+requests
import java.util.function.Consumer; | ||
|
||
/** A {@link KeyedCoProcessOperator} that supports holding back watermarks with a static delay. */ | ||
public class AsyncKeyedCoProcessOperatorWithWatermarkDelay<K, IN1, IN2, OUT> |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
I cant see this class called anywhere. Even from test code.
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
You're right, It will be called at StreamExecIntervalJoin
in the future after we implement async version rowJoinFunc
.
3640756
to
7ddeb5d
Compare
...org/apache/flink/runtime/asyncprocessing/functions/DeclaringAsyncKeyedCoProcessFunction.java
Outdated
Show resolved
Hide resolved
.../java/org/apache/flink/runtime/asyncprocessing/operators/co/AsyncKeyedCoProcessOperator.java
Show resolved
Hide resolved
...link/runtime/asyncprocessing/operators/co/AsyncKeyedCoProcessOperatorWithWatermarkDelay.java
Outdated
Show resolved
Hide resolved
@@ -387,8 +387,11 @@ public Watermark preProcessWatermark(Watermark watermark) throws Exception { | |||
* perform async state here. Only some synchronous logic is suggested. | |||
* | |||
* @param watermark the advanced watermark. | |||
* @return the watermark that should be emitted to downstream. |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
We should allow returning null
if there is no need emitting any watermark downstream.
And please update the javadoc of this method explaining this.
...ache/flink/table/runtime/operators/join/interval/asyncprocess/AsyncProcTimeIntervalJoin.java
Outdated
Show resolved
Hide resolved
rightCache.asyncRemove(key); | ||
} | ||
}); | ||
return emitResult.get(); |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
We better not blocking get()
here. It will block the task thread as well as those callbacks for async state requests.
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
👍I changed the return value to StateFuture<Boolean>
.
...c/main/java/org/apache/flink/table/runtime/operators/join/interval/SyncTimeIntervalJoin.java
Outdated
Show resolved
Hide resolved
3351542
to
1c0dca7
Compare
joinFunction.setJoinKey(ctx.getCurrentKey()); | ||
joinCollector.setInnerCollector(out); |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
the internal state of joinFunction
and joinCollector
should be maintained by DeclaredVariable
since it belongs to the context for async processing.
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
👍 Good suggestion, I rewrote AsyncTimeIntervalJoin
with declareChain
.
...g/apache/flink/table/runtime/operators/join/interval/asyncprocess/AsyncTimeIntervalJoin.java
Outdated
Show resolved
Hide resolved
...g/apache/flink/table/runtime/operators/join/interval/asyncprocess/AsyncTimeIntervalJoin.java
Outdated
Show resolved
Hide resolved
throw new IllegalAccessException("This method is replaced by declareProcess1."); | ||
} | ||
|
||
/** Override this method or use {@link #declareProcess2} instead. */ |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
Actually we dont recommend to use this method. How about rewrite this javadoc?
/** Override this method or use {@link #declareProcess2} instead. */ | |
/** Override and finalize this method. Please use {@link #declareProcess2} instead. */ |
*/ | ||
public void postProcessWatermark(Watermark watermark) throws Exception {} | ||
public Watermark postProcessWatermark(Watermark watermark) throws Exception { |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
How about change the javadoc of this method to :
A hook that will be invoked after finishing advancing the watermark and right before the watermark being emitting downstream. Here is a chance for customization of the emitting watermark. ....(and following description)
@@ -417,12 +419,12 @@ public <X> void output(OutputTag<X> outputTag, X value) { | |||
} | |||
|
|||
@VisibleForTesting | |||
MapState<Long, List<IntervalJoinOperator.BufferEntry<T1>>> getLeftBuffer() { | |||
public MapState<Long, List<IntervalJoinOperator.BufferEntry<T1>>> getLeftBuffer() { |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
Seems unnecessary?
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
Because AsyncIntervalJoinOperator
is moved into asyncprocessing
package, and AsyncIntervalJoinOperatorTest
still needs these methods.
flink-runtime/src/main/java/org/apache/flink/streaming/api/datastream/ConnectedStreams.java
Show resolved
Hide resolved
.../main/java/org/apache/flink/table/planner/plan/nodes/exec/stream/StreamExecIntervalJoin.java
Outdated
Show resolved
Hide resolved
flink-runtime/src/main/java/org/apache/flink/streaming/api/datastream/ConnectedStreams.java
Outdated
Show resolved
Hide resolved
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
Thanks for the update! LGTM. Would you please update the description of this PR and remove the unrelated change?
What is the purpose of the change
This PR introduces
AsyncKeyedCoProcessOperator
Brief change log
AsyncKeyedCoProcessOperator
AsyncKeyedCoProcessOperatorWithWatermarkDelay
DeclaringAsyncKeyedCoProcessFunction
AsyncIntervalJoinOperator
too.a.f.runtime.asyncprocessing.operators.co
package.Verifying this change
AsyncKeyedCoProcessOperatorTest
Does this pull request potentially affect one of the following parts:
@Public(Evolving)
: (no)Documentation