WencongLiu commented on code in PR #23362: URL: https://github.com/apache/flink/pull/23362#discussion_r1378416760
########## docs/content/docs/dev/datastream/dataset_migration.md: ########## @@ -0,0 +1,778 @@ +--- +title: "How to Migrate from DataSet to DataStream" +weight: 302 +type: docs +--- +<!-- +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. +--> + +# How to Migrate from DataSet to DataStream + +The DataSet API has been formally deprecated and will no longer receive active maintenance and support. It will be removed in the +Flink 2.0 version. Flink users are recommended to migrate from the DataSet API to the DataStream API, Table API and SQL for their +data processing requirements. + +To build the same data processing application, the DataSet APIs can be divided into four categories when migrating them to +DataStream APIs. + +Category 1: These DataSet APIs can be migrated to DataStream APIs with same semantic and same processing behavior. + +Category 2: These DataSet APIs can be migrated to DataStream APIs with different semantic but same processing behavior. This will +make the job code more complex. + +Category 3: These DataSet APIs can be migrated to DataStream APIs with different semantic and different processing behavior. This +will involve additional computation and I/O costs. + +Category 4: These DataSet APIs are not supported by DataStream APIs. + +The subsequent sections will first introduce how to set the execution environment and source/sink, then provide detailed explanations on how to migrate +each category of DataSet APIs to the DataStream APIs, highlighting the specific considerations and challenges associated with each +category. + + +## Setting the execution environment + +To execute a DataSet pipeline by DataStream API, we should first start by replacing `ExecutionEnvironment` with `StreamExecutionEnvironment`. + +<table class="table table-bordered"> + <thead> + <tr> + <th class="text-left">DataSet</th> + <th class="text-left">DataStream</th> + </tr> + </thead> + <tbody> + <tr> + <td> + {{< highlight "java" >}} +// Create the execution environment +ExecutionEnvironment.getExecutionEnvironment(); +// Create the local execution environment +ExecutionEnvironment.createLocalEnvironment(); +// Create the collection environment +new CollectionEnvironment(); +// Create the remote environment +ExecutionEnvironment.createRemoteEnvironment(String host, int port, String... jarFiles); + {{< /highlight >}} + </td> + <td> + {{< highlight "java" >}} +// Create the execution environment +StreamExecutionEnvironment.getExecutionEnvironment(); +// Create the local execution environment +StreamExecutionEnvironment.createLocalEnvironment(); +// The collection environment is not supported. +// Create the remote environment +StreamExecutionEnvironment.createRemoteEnvironment(String host, int port, String... jarFiles); + {{< /highlight >}} + </td> + </tr> + </tbody> +</table> + +As the source of DataSet is always bounded, the `RuntimeMode` must be set to `RuntimeMode.BATCH` to make Flink execute in batch mode. + +```java +StreamExecutionEnvironment executionEnvironment = // [...]; +executionEnvironment.setRuntimeMode(RuntimeExecutionMode.BATCH); +``` + +## Using the streaming sources and sinks + +### Sources + +The DataStream API uses `DataStreamSource` to read records from external system, while the DataSet API uses the `DataSource`. + +<table class="table table-bordered"> + <thead> + <tr> + <th class="text-left">DataSet</th> + <th class="text-left">DataStream</th> + </tr> + </thead> + <tbody> + <tr> + <td> + {{< highlight "java" >}} +// Read data from file +DataSource<> source = ExecutionEnvironment.readFile(inputFormat, filePath); +// Read data from collection +DataSource<> source = ExecutionEnvironment.fromCollection(data); +// Read data from inputformat +DataSource<> source = ExecutionEnvironment.createInput(inputFormat) + {{< /highlight >}} + </td> + <td> + {{< highlight "java" >}} +// Read data from file +DataStreamSource<> source = StreamExecutionEnvironment.readFile(inputFormat, filePath); +// Read data from collection +DataStreamSource<> source = StreamExecutionEnvironment.fromCollection(data); +// Read data from inputformat +DataStreamSource<> source = StreamExecutionEnvironment.createInput(inputFormat) + {{< /highlight >}} + </td> + </tr> + </tbody> +</table> + +### Sinks + +The DataStream API uses `DataStreamSink` to write records to external system, while the +DataSet API uses the `DataSink`. + +<table class="table table-bordered"> + <thead> + <tr> + <th class="text-left">DataSet</th> + <th class="text-left">DataStream</th> + </tr> + </thead> + <tbody> + <tr> + <td> + {{< highlight "java" >}} +// Write to outputformat +DataSink<> sink = dataSet.output(outputFormat); +// Write to csv file +DataSink<> sink = dataSet.writeAsCsv(filePath); +// Write to text file +DataSink<> sink = dataSet.writeAsText(filePath); + {{< /highlight >}} + </td> + <td> + {{< highlight "java" >}} +// Write to sink function or sink +DataStreamSink<> sink = dataStream.addSink(sinkFunction) +DataStreamSink<> sink = dataStream.sinkTo(sink) +// Write to csv file +DataStreamSink<> sink = dataStream.writeAsCsv(path); +// Write to text file +DataStreamSink<> sink = dataStream.writeAsText(path); + {{< /highlight >}} + </td> + </tr> + </tbody> +</table> + +If you are looking for pre-defined source and sink connectors of DataStream, please check the [Connector Docs]({{< ref "docs/connectors/datastream/overview" >}}) + +## Migrating DataSet APIs + +### Category 1 + +For Category 1, these DataSet APIs can be migrated to DataStream APIs with same semantic and same processing behavior. This means the migration is +relatively straightforward and does not require significant modifications or complexity in the job code. + +<table class="table table-bordered"> + <thead> + <tr> + <th class="text-left">Operations</th> + <th class="text-left">DataSet</th> + <th class="text-left">DataStream</th> + </tr> + </thead> + <tbody> + <tr> + <td>Map</td> + <td> + {{< highlight "java" >}} +dataSet.map(new MapFunction<>(){ +// implement user-defined map logic +}); + {{< /highlight >}} + </td> + <td> + {{< highlight "java" >}} +dataStream.map(new MapFunction<>(){ +// implement user-defined map logic +}); + {{< /highlight >}} + </td> + </tr> + <tr> + <td>FlatMap</td> + <td> + {{< highlight "java" >}} +dataSet.flatMap(new FlatMapFunction<>(){ +// implement user-defined flatmap logic +}); + {{< /highlight >}} + </td> + <td> + {{< highlight "java" >}} +dataStream.flatMap(new FlatMapFunction<>(){ +// implement user-defined flatmap logic +}); + {{< /highlight >}} + </td> + </tr> + <tr> + <td>Filter</td> + <td> + {{< highlight "java" >}} +dataSet.filter(new FilterFunction<>(){ +// implement user-defined filter logic +}); + {{< /highlight >}} + </td> + <td> + {{< highlight "java" >}} +dataStream.filter(new FilterFunction<>(){ +// implement user-defined filter logic +}); + {{< /highlight >}} + </td> + </tr> + <tr> + <td>Union</td> + <td> + {{< highlight "java" >}} +dataSet1.union(dataSet2); + {{< /highlight >}} + </td> + <td> + {{< highlight "java" >}} +dataStream1.union(dataStream2); + {{< /highlight >}} + </td> + </tr> + <tr> + <td>Rebalance</td> + <td> + {{< highlight "java" >}} +dataSet.rebalance(); + {{< /highlight >}} + </td> + <td> + {{< highlight "java" >}} +dataStream.rebalance(); + {{< /highlight >}} + </td> + </tr> + <tr> + <td>Project</td> + <td> + {{< highlight "java" >}} +DataSet<Tuple3<>> dataSet = // [...] +dataSet.project(2,0); + {{< /highlight >}} + </td> + <td> + {{< highlight "java" >}} +DataStream<Tuple3<>> dataStream = // [...] +dataStream.project(2,0); + {{< /highlight >}} + </td> + </tr> + <tr> + <td>Reduce on Grouped DataSet</td> + <td> + {{< highlight "java" >}} +DataSet<Tuple2<>> dataSet = // [...] +dataSet.groupBy(value -> value.f0) + .reduce(new ReduceFunction<>(){ + // implement user-defined reduce logic + }); + {{< /highlight >}} + </td> + <td> + {{< highlight "java" >}} +DataStream<Tuple2<>> dataStream = // [...] +dataStream.keyBy(value -> value.f0) + .reduce(new ReduceFunction<>(){ + // implement user-defined reduce logic + }); + {{< /highlight >}} + </td> + </tr> + <tr> + <td>Aggregate on Grouped DataSet</td> + <td> + {{< highlight "java" >}} +DataSet<Tuple2<>> dataSet = // [...] +// compute sum of the second field +dataSet.groupBy(value -> value.f0) + .aggregate(SUM, 1); +// compute min of the second field +dataSet.groupBy(value -> value.f0) + .aggregate(MIN, 1); +// compute max of the second field +dataSet.groupBy(value -> value.f0) + .aggregate(MAX, 1); + {{< /highlight >}} + </td> + <td> + {{< highlight "java" >}} +DataStream<Tuple2<>> dataStream = // [...] +// compute sum of the second field +dataStream.keyBy(value -> value.f0) + .sum(1); +// compute min of the second field +dataStream.keyBy(value -> value.f0) + .min(1); +// compute max of the second field +dataStream.keyBy(value -> value.f0) + .max(1); + {{< /highlight >}} + </td> + </tr> + </tbody> +</table> + +### Category 2 + +For category 2, these DataSet APIs can be migrated to DataStream APIs with different semantic but same processing behavior. +The different semantic will introduce additional complexity of job codes and require developers more migration efforts. + +<table class="table table-bordered"> + <thead> + <tr> + <th class="text-left">Operations</th> + <th class="text-left">DataSet</th> + <th class="text-left">DataStream</th> + </tr> + </thead> + <tbody> + <tr> + <td>Distinct</td> + <td> + {{< highlight "java" >}} +DataSet<Integer> dataSet = // [...] +dataSet.distinct(); + {{< /highlight >}} + </td> + <td> + {{< highlight "java" >}} +DataStream<Integer> dataStream = // [...] +dataStream.keyBy(value -> value) + .reduce((value1, value2) -> value1); + {{< /highlight >}} + </td> + </tr> + <tr> + <td>Hash-Partition</td> + <td> + {{< highlight "java" >}} +DataSet<Tuple2<>> dataSet = // [...] +dataSet.partitionByHash(value -> value.f0); + {{< /highlight >}} + </td> + <td> + {{< highlight "java" >}} +DataStream<Tuple2<>> dataStream = // [...] +// partition by the hashcode of key +dataStream.partitionCustom( + (key, numSubpartition) -> key.hashCode() % numSubpartition, + value -> value.f0); + {{< /highlight >}} + </td> + </tr> + </tbody> +</table> + +Operations on a full DataSet correspond to the global window aggregation in DataStream with a custom `EndOfStreamWindows` +WindowAssigner that invokes the computation of window at the end of the inputs. Below is an example code snippet of +`EndOfStreamWindows` that will be reused in the rest of this document. Review Comment: Done. -- 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: [email protected] For queries about this service, please contact Infrastructure at: [email protected]
