[
https://issues.apache.org/jira/browse/FLINK-5026?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=15701724#comment-15701724
]
ASF GitHub Bot commented on FLINK-5026:
---------------------------------------
Github user kl0u commented on a diff in the pull request:
https://github.com/apache/flink/pull/2788#discussion_r89759655
--- Diff:
flink-streaming-java/src/main/java/org/apache/flink/streaming/api/datastream/ConnectedStreams.java
---
@@ -234,62 +235,62 @@ public StreamExecutionEnvironment
getExecutionEnvironment() {
}
/**
- * Applies the given {@link TimelyCoFlatMapFunction} on the connected
input streams,
+ * Applies the given {@link CoProcessFunction} on the connected input
streams,
* thereby creating a transformed output stream.
*
* <p>The function will be called for every element in the streams and
can produce
* zero or more output. The function can also query the time and set
timers. When
* reacting to the firing of set timers the function can emit yet more
elements.
*
- * <p>A {@link
org.apache.flink.streaming.api.functions.co.RichTimelyCoFlatMapFunction}
+ * <p>A {@link RichCoProcessFunction}
* can be used to gain access to features provided by the
* {@link org.apache.flink.api.common.functions.RichFunction} interface.
*
- * @param coFlatMapper The {@link TimelyCoFlatMapFunction} that is
called for each element
+ * @param coProcessFunction The {@link CoProcessFunction} that is
called for each element
* in the stream.
*
- * @param <R> The of elements emitted by the {@code
TimelyCoFlatMapFunction}.
+ * @param <R> The of elements emitted by the {@code CoProcessFunction}.
*
* @return The transformed {@link DataStream}.
*/
- public <R> SingleOutputStreamOperator<R> flatMap(
- TimelyCoFlatMapFunction<IN1, IN2, R> coFlatMapper) {
+ public <R> SingleOutputStreamOperator<R> process(
+ CoProcessFunction<IN1, IN2, R> coProcessFunction) {
- TypeInformation<R> outTypeInfo =
TypeExtractor.getBinaryOperatorReturnType(coFlatMapper,
- TimelyCoFlatMapFunction.class, false, true,
getType1(), getType2(),
+ TypeInformation<R> outTypeInfo =
TypeExtractor.getBinaryOperatorReturnType(coProcessFunction,
+ CoProcessFunction.class, false, true,
getType1(), getType2(),
Utils.getCallLocationName(), true);
- return flatMap(coFlatMapper, outTypeInfo);
+ return process(coProcessFunction, outTypeInfo);
}
/**
- * Applies the given {@link TimelyCoFlatMapFunction} on the connected
input streams,
+ * Applies the given {@link CoProcessFunction} on the connected input
streams,
* thereby creating a transformed output stream.
*
* <p>The function will be called for every element in the streams and
can produce
* zero or more output. The function can also query the time and set
timers. When
* reacting to the firing of set timers the function can emit yet more
elements.
*
- * <p>A {@link
org.apache.flink.streaming.api.functions.co.RichTimelyCoFlatMapFunction}
+ * <p>A {@link RichCoProcessFunction}
* can be used to gain access to features provided by the
* {@link org.apache.flink.api.common.functions.RichFunction} interface.
*
- * @param coFlatMapper The {@link TimelyCoFlatMapFunction} that is
called for each element
+ * @param coProcessFunction The {@link CoProcessFunction} that is
called for each element
* in the stream.
*
- * @param <R> The of elements emitted by the {@code
TimelyCoFlatMapFunction}.
+ * @param <R> The of elements emitted by the {@code CoProcessFunction}.
*
--- End diff --
<R> The type of elements emitted by the {@code CoProcessFunction}.
> Rename TimelyFlatMap to Process
> -------------------------------
>
> Key: FLINK-5026
> URL: https://issues.apache.org/jira/browse/FLINK-5026
> Project: Flink
> Issue Type: Improvement
> Components: Streaming
> Reporter: Aljoscha Krettek
> Assignee: Aljoscha Krettek
> Fix For: 1.2.0
>
>
> The method on {{KeyedDataStream}} would be called {{process()}} and the
> function itself would be called {{ProcessFunction}}.
> The reason for this is that {{TimelyFlatMapFunction}} is a bit of a mouthful
> and with the additions to the timer API and state the {{ProcessFunction}}
> could become the basic, low-level, user-facing API for cases where users
> nowadays implement their own operator.
--
This message was sent by Atlassian JIRA
(v6.3.4#6332)