From 568dff123d7ede05185be60842530076e234b3fa Mon Sep 17 00:00:00 2001 From: Aljoscha Krettek Date: Wed, 3 Sep 2014 14:50:42 +0200 Subject: [PATCH] Move RichFunctions to api.common package They were in api.java before but they can be used from Scala, just like the regular functions. --- .../flink/api/avro/AvroOutputFormatTest.java | 2 +- .../avro/testjar/AvroExternalJarProgram.java | 4 ++-- .../mapred/example/WordCount.java | 4 ++-- .../mapreduce/example/WordCount.java | 4 ++-- .../spargel/java/VertexCentricIteration.java | 2 +- .../SpargelPageRankCountingVertices.java | 2 +- .../SpargelConnectedComponentsITCase.java | 2 +- .../connectors/json/JSONParseFlatMap.java | 2 +- .../api/datastream/BatchedDataStream.java | 4 ++-- .../api/datastream/ConnectedDataStream.java | 2 +- .../streaming/api/datastream/DataStream.java | 8 +++---- .../api/datastream/GroupedDataStream.java | 2 +- .../flink/streaming/api/IterateTest.java | 2 +- .../operator/GroupedBatchGroupReduceTest.java | 2 +- .../streamcomponent/StreamComponentTest.java | 2 +- .../serialization/TypeSerializationTest.java | 2 +- .../ml/IncrementalLearningSkeleton.java | 3 +++ .../flink/client/testjar/WordCount.java | 2 +- .../compiler/BranchingPlansCompilerTest.java | 2 +- .../CachedMatchStrategyCompilerTest.java | 2 +- .../compiler/CoGroupSolutionSetFirstTest.java | 4 ++-- .../compiler/GroupReduceCompilationTest.java | 2 +- .../compiler/IterationsCompilerTest.java | 8 +++---- .../flink/compiler/ReduceCompilationTest.java | 2 +- .../UnionPropertyPropagationTest.java | 2 +- .../WorksetIterationsJavaApiCompilerTest.java | 6 ++--- .../testfunctions/DummyFlatJoinFunction.java | 2 +- .../testfunctions/IdentityGroupReducer.java | 4 ++-- .../testfunctions/IdentityMapper.java | 2 +- .../testfunctions/SelectOneReducer.java | 2 +- .../testfunctions/Top1GroupReducer.java | 4 ++-- .../functions/RichCoGroupFunction.java | 2 +- .../common}/functions/RichCrossFunction.java | 2 +- .../common}/functions/RichFilterFunction.java | 2 +- .../functions/RichFlatCombineFunction.java | 2 +- .../functions/RichFlatJoinFunction.java | 2 +- .../functions/RichFlatMapFunction.java | 2 +- .../functions/RichGroupReduceFunction.java | 6 +---- .../common}/functions/RichJoinFunction.java | 2 +- .../common}/functions/RichMapFunction.java | 2 +- .../functions/RichMapPartitionFunction.java | 2 +- .../common}/functions/RichReduceFunction.java | 2 +- .../flink/example/java/clustering/KMeans.java | 2 +- .../example/java/ml/LinearRegression.java | 2 +- .../EmptyFieldsCountAccumulator.java | 2 +- .../org/apache/flink/api/java/DataSet.java | 22 +++++++++---------- .../api/java/functions/FlatMapIterator.java | 3 ++- .../java/functions/FunctionAnnotation.java | 16 +++++++------- .../java/functions/GroupReduceIterator.java | 1 + .../api/java/operators/AggregateOperator.java | 4 ++-- .../api/java/operators/CoGroupOperator.java | 6 ++--- .../api/java/operators/DistinctOperator.java | 4 ++-- .../java/operators/GroupReduceOperator.java | 2 +- .../api/java/operators/JoinOperator.java | 8 +++---- .../operators/SingleInputUdfOperator.java | 4 ++-- .../api/java/operators/SortedGrouping.java | 4 ++-- .../java/operators/TwoInputUdfOperator.java | 4 ++-- .../flink/api/java/operators/UdfOperator.java | 4 ++-- .../api/java/operators/UnsortedGrouping.java | 8 +++---- .../translation/KeyExtractingMapper.java | 2 +- .../translation/KeyRemovingMapper.java | 2 +- .../PlanUnwrappingReduceGroupOperator.java | 2 +- .../SemanticPropertiesTranslationTest.java | 2 ++ .../DeltaIterationTranslationTest.java | 6 ++--- .../translation/ReduceTranslationTests.java | 2 +- .../type/extractor/TypeExtractorTest.java | 12 +++++----- .../lambdas/LambdaExtractionTest.java | 2 +- .../operators/CachedMatchTaskTest.java | 5 +---- .../operators/CoGroupTaskExternalITCase.java | 4 +--- .../runtime/operators/CoGroupTaskTest.java | 4 +--- .../operators/CombineTaskExternalITCase.java | 2 +- .../runtime/operators/CombineTaskTest.java | 4 +--- .../operators/ReduceTaskExternalITCase.java | 4 +--- .../runtime/operators/ReduceTaskTest.java | 4 +--- .../operators/chaining/ChainTaskTest.java | 2 +- .../drivers/AllGroupReduceDriverTest.java | 2 +- .../drivers/AllReduceDriverTest.java | 2 +- .../drivers/GroupReduceDriverTest.java | 2 +- .../drivers/ReduceCombineDriverTest.java | 2 +- .../operators/drivers/ReduceDriverTest.java | 2 +- .../CombiningUnilateralSortMergerITCase.java | 2 +- ...tipleJoinsWithSolutionSetCompilerTest.java | 6 ++--- .../BulkIterationWithAllReducerITCase.java | 2 +- ...oGroupConnectedComponentsSecondITCase.java | 6 ++--- .../DependencyConnectedComponentsITCase.java | 6 ++--- .../aggregators/AggregatorsITCase.java | 4 ++-- ...ntsWithParametrizableAggregatorITCase.java | 4 ++-- ...tsWithParametrizableConvergenceITCase.java | 6 ++--- .../test/javaApiOperators/CoGroupITCase.java | 2 +- .../test/javaApiOperators/CrossITCase.java | 2 +- .../test/javaApiOperators/DistinctITCase.java | 2 +- .../test/javaApiOperators/FilterITCase.java | 2 +- .../test/javaApiOperators/FlatMapITCase.java | 2 +- .../javaApiOperators/GroupReduceITCase.java | 4 ++-- .../test/javaApiOperators/JoinITCase.java | 2 +- .../test/javaApiOperators/MapITCase.java | 2 +- .../test/javaApiOperators/ReduceITCase.java | 2 +- .../test/javaApiOperators/UnionITCase.java | 2 +- .../test/util/testjar/KMeansForTest.java | 4 ++-- 99 files changed, 168 insertions(+), 178 deletions(-) rename {flink-java/src/main/java/org/apache/flink/api/java => flink-core/src/main/java/org/apache/flink/api/common}/functions/RichCoGroupFunction.java (97%) rename {flink-java/src/main/java/org/apache/flink/api/java => flink-core/src/main/java/org/apache/flink/api/common}/functions/RichCrossFunction.java (97%) rename {flink-java/src/main/java/org/apache/flink/api/java => flink-core/src/main/java/org/apache/flink/api/common}/functions/RichFilterFunction.java (97%) rename {flink-java/src/main/java/org/apache/flink/api/java => flink-core/src/main/java/org/apache/flink/api/common}/functions/RichFlatCombineFunction.java (97%) rename {flink-java/src/main/java/org/apache/flink/api/java => flink-core/src/main/java/org/apache/flink/api/common}/functions/RichFlatJoinFunction.java (97%) rename {flink-java/src/main/java/org/apache/flink/api/java => flink-core/src/main/java/org/apache/flink/api/common}/functions/RichFlatMapFunction.java (97%) rename {flink-java/src/main/java/org/apache/flink/api/java => flink-core/src/main/java/org/apache/flink/api/common}/functions/RichGroupReduceFunction.java (93%) rename {flink-java/src/main/java/org/apache/flink/api/java => flink-core/src/main/java/org/apache/flink/api/common}/functions/RichJoinFunction.java (97%) rename {flink-java/src/main/java/org/apache/flink/api/java => flink-core/src/main/java/org/apache/flink/api/common}/functions/RichMapFunction.java (97%) rename {flink-java/src/main/java/org/apache/flink/api/java => flink-core/src/main/java/org/apache/flink/api/common}/functions/RichMapPartitionFunction.java (97%) rename {flink-java/src/main/java/org/apache/flink/api/java => flink-core/src/main/java/org/apache/flink/api/common}/functions/RichReduceFunction.java (97%) diff --git a/flink-addons/flink-avro/src/test/java/org/apache/flink/api/avro/AvroOutputFormatTest.java b/flink-addons/flink-avro/src/test/java/org/apache/flink/api/avro/AvroOutputFormatTest.java index 386f318d37b..7ae8df07f00 100644 --- a/flink-addons/flink-avro/src/test/java/org/apache/flink/api/avro/AvroOutputFormatTest.java +++ b/flink-addons/flink-avro/src/test/java/org/apache/flink/api/avro/AvroOutputFormatTest.java @@ -30,7 +30,7 @@ import org.apache.avro.reflect.ReflectDatumReader; import org.apache.avro.specific.SpecificDatumReader; import org.apache.flink.api.java.DataSet; import org.apache.flink.api.java.ExecutionEnvironment; -import org.apache.flink.api.java.functions.RichMapFunction; +import org.apache.flink.api.common.functions.RichMapFunction; import org.apache.flink.api.java.io.AvroOutputFormat; import org.apache.flink.api.java.record.io.avro.example.User; import org.apache.flink.api.java.tuple.Tuple3; diff --git a/flink-addons/flink-avro/src/test/java/org/apache/flink/api/avro/testjar/AvroExternalJarProgram.java b/flink-addons/flink-avro/src/test/java/org/apache/flink/api/avro/testjar/AvroExternalJarProgram.java index 75b7da69336..d0e52fdc230 100644 --- a/flink-addons/flink-avro/src/test/java/org/apache/flink/api/avro/testjar/AvroExternalJarProgram.java +++ b/flink-addons/flink-avro/src/test/java/org/apache/flink/api/avro/testjar/AvroExternalJarProgram.java @@ -40,8 +40,8 @@ import org.apache.avro.io.DatumWriter; import org.apache.avro.reflect.ReflectData; import org.apache.avro.reflect.ReflectDatumWriter; import org.apache.flink.api.avro.AvroBaseValue; -import org.apache.flink.api.java.functions.RichMapFunction; -import org.apache.flink.api.java.functions.RichReduceFunction; +import org.apache.flink.api.common.functions.RichMapFunction; +import org.apache.flink.api.common.functions.RichReduceFunction; import org.apache.flink.api.java.tuple.Tuple2; import org.apache.flink.api.java.DataSet; import org.apache.flink.api.java.ExecutionEnvironment; diff --git a/flink-addons/flink-hadoop-compatibility/src/main/java/org/apache/flink/hadoopcompatibility/mapred/example/WordCount.java b/flink-addons/flink-hadoop-compatibility/src/main/java/org/apache/flink/hadoopcompatibility/mapred/example/WordCount.java index ba09e77b95d..623b58d18e0 100644 --- a/flink-addons/flink-hadoop-compatibility/src/main/java/org/apache/flink/hadoopcompatibility/mapred/example/WordCount.java +++ b/flink-addons/flink-hadoop-compatibility/src/main/java/org/apache/flink/hadoopcompatibility/mapred/example/WordCount.java @@ -18,7 +18,7 @@ package org.apache.flink.hadoopcompatibility.mapred.example; -import org.apache.flink.api.java.functions.RichMapFunction; +import org.apache.flink.api.common.functions.RichMapFunction; import org.apache.hadoop.fs.Path; import org.apache.hadoop.io.IntWritable; import org.apache.hadoop.io.LongWritable; @@ -29,7 +29,7 @@ import org.apache.hadoop.mapred.TextOutputFormat; import org.apache.flink.api.java.DataSet; import org.apache.flink.api.java.ExecutionEnvironment; import org.apache.flink.api.java.aggregation.Aggregations; -import org.apache.flink.api.java.functions.RichFlatMapFunction; +import org.apache.flink.api.common.functions.RichFlatMapFunction; import org.apache.flink.api.java.tuple.Tuple2; import org.apache.flink.hadoopcompatibility.mapred.HadoopInputFormat; import org.apache.flink.hadoopcompatibility.mapred.HadoopOutputFormat; diff --git a/flink-addons/flink-hadoop-compatibility/src/main/java/org/apache/flink/hadoopcompatibility/mapreduce/example/WordCount.java b/flink-addons/flink-hadoop-compatibility/src/main/java/org/apache/flink/hadoopcompatibility/mapreduce/example/WordCount.java index c00a14ae2f0..33ab97cc4ea 100644 --- a/flink-addons/flink-hadoop-compatibility/src/main/java/org/apache/flink/hadoopcompatibility/mapreduce/example/WordCount.java +++ b/flink-addons/flink-hadoop-compatibility/src/main/java/org/apache/flink/hadoopcompatibility/mapreduce/example/WordCount.java @@ -19,8 +19,8 @@ package org.apache.flink.hadoopcompatibility.mapreduce.example; import org.apache.flink.api.java.aggregation.Aggregations; -import org.apache.flink.api.java.functions.RichFlatMapFunction; -import org.apache.flink.api.java.functions.RichMapFunction; +import org.apache.flink.api.common.functions.RichFlatMapFunction; +import org.apache.flink.api.common.functions.RichMapFunction; import org.apache.flink.api.java.tuple.Tuple2; import org.apache.flink.util.Collector; import org.apache.hadoop.fs.Path; diff --git a/flink-addons/flink-spargel/src/main/java/org/apache/flink/spargel/java/VertexCentricIteration.java b/flink-addons/flink-spargel/src/main/java/org/apache/flink/spargel/java/VertexCentricIteration.java index 777cf9d00b1..37cc549ee1f 100644 --- a/flink-addons/flink-spargel/src/main/java/org/apache/flink/spargel/java/VertexCentricIteration.java +++ b/flink-addons/flink-spargel/src/main/java/org/apache/flink/spargel/java/VertexCentricIteration.java @@ -29,7 +29,7 @@ import org.apache.commons.lang3.Validate; import org.apache.flink.api.common.aggregators.Aggregator; import org.apache.flink.api.java.DataSet; import org.apache.flink.api.java.operators.DeltaIteration; -import org.apache.flink.api.java.functions.RichCoGroupFunction; +import org.apache.flink.api.common.functions.RichCoGroupFunction; import org.apache.flink.api.java.operators.CoGroupOperator; import org.apache.flink.api.java.operators.CustomUnaryOperation; import org.apache.flink.api.java.tuple.Tuple2; diff --git a/flink-addons/flink-spargel/src/main/java/org/apache/flink/spargel/java/examples/SpargelPageRankCountingVertices.java b/flink-addons/flink-spargel/src/main/java/org/apache/flink/spargel/java/examples/SpargelPageRankCountingVertices.java index 01d2cd70d71..94946ef2a96 100644 --- a/flink-addons/flink-spargel/src/main/java/org/apache/flink/spargel/java/examples/SpargelPageRankCountingVertices.java +++ b/flink-addons/flink-spargel/src/main/java/org/apache/flink/spargel/java/examples/SpargelPageRankCountingVertices.java @@ -21,7 +21,7 @@ package org.apache.flink.spargel.java.examples; import org.apache.flink.api.common.functions.FlatMapFunction; import org.apache.flink.api.common.functions.MapFunction; import org.apache.flink.api.common.functions.ReduceFunction; -import org.apache.flink.api.java.functions.RichMapFunction; +import org.apache.flink.api.common.functions.RichMapFunction; import org.apache.flink.api.java.tuple.Tuple2; import org.apache.flink.api.java.tuple.Tuple3; import org.apache.flink.configuration.Configuration; diff --git a/flink-addons/flink-spargel/src/test/java/org/apache/flink/test/spargel/SpargelConnectedComponentsITCase.java b/flink-addons/flink-spargel/src/test/java/org/apache/flink/test/spargel/SpargelConnectedComponentsITCase.java index 16b004c6b9f..948a7087ce4 100644 --- a/flink-addons/flink-spargel/src/test/java/org/apache/flink/test/spargel/SpargelConnectedComponentsITCase.java +++ b/flink-addons/flink-spargel/src/test/java/org/apache/flink/test/spargel/SpargelConnectedComponentsITCase.java @@ -21,7 +21,7 @@ package org.apache.flink.test.spargel; import java.io.BufferedReader; -import org.apache.flink.api.java.functions.RichMapFunction; +import org.apache.flink.api.common.functions.RichMapFunction; import org.apache.flink.api.java.tuple.Tuple2; import org.apache.flink.api.java.DataSet; import org.apache.flink.api.java.ExecutionEnvironment; diff --git a/flink-addons/flink-streaming/flink-streaming-connectors/src/main/java/org/apache/flink/streaming/connectors/json/JSONParseFlatMap.java b/flink-addons/flink-streaming/flink-streaming-connectors/src/main/java/org/apache/flink/streaming/connectors/json/JSONParseFlatMap.java index 96b1bf75a76..64f22ec5024 100644 --- a/flink-addons/flink-streaming/flink-streaming-connectors/src/main/java/org/apache/flink/streaming/connectors/json/JSONParseFlatMap.java +++ b/flink-addons/flink-streaming/flink-streaming-connectors/src/main/java/org/apache/flink/streaming/connectors/json/JSONParseFlatMap.java @@ -17,7 +17,7 @@ package org.apache.flink.streaming.connectors.json; -import org.apache.flink.api.java.functions.RichFlatMapFunction; +import org.apache.flink.api.common.functions.RichFlatMapFunction; import org.apache.sling.commons.json.JSONException; /** diff --git a/flink-addons/flink-streaming/flink-streaming-core/src/main/java/org/apache/flink/streaming/api/datastream/BatchedDataStream.java b/flink-addons/flink-streaming/flink-streaming-core/src/main/java/org/apache/flink/streaming/api/datastream/BatchedDataStream.java index 0249a1f12aa..2565ce1de30 100755 --- a/flink-addons/flink-streaming/flink-streaming-core/src/main/java/org/apache/flink/streaming/api/datastream/BatchedDataStream.java +++ b/flink-addons/flink-streaming/flink-streaming-core/src/main/java/org/apache/flink/streaming/api/datastream/BatchedDataStream.java @@ -19,8 +19,8 @@ package org.apache.flink.streaming.api.datastream; import org.apache.flink.api.common.functions.GroupReduceFunction; import org.apache.flink.api.common.functions.ReduceFunction; -import org.apache.flink.api.java.functions.RichGroupReduceFunction; -import org.apache.flink.api.java.functions.RichReduceFunction; +import org.apache.flink.api.common.functions.RichGroupReduceFunction; +import org.apache.flink.api.common.functions.RichReduceFunction; import org.apache.flink.streaming.api.function.aggregation.AggregationFunction; import org.apache.flink.streaming.api.function.aggregation.MaxAggregationFunction; import org.apache.flink.streaming.api.function.aggregation.MinAggregationFunction; diff --git a/flink-addons/flink-streaming/flink-streaming-core/src/main/java/org/apache/flink/streaming/api/datastream/ConnectedDataStream.java b/flink-addons/flink-streaming/flink-streaming-core/src/main/java/org/apache/flink/streaming/api/datastream/ConnectedDataStream.java index 9529dcdc51d..6108eecc3a8 100755 --- a/flink-addons/flink-streaming/flink-streaming-core/src/main/java/org/apache/flink/streaming/api/datastream/ConnectedDataStream.java +++ b/flink-addons/flink-streaming/flink-streaming-core/src/main/java/org/apache/flink/streaming/api/datastream/ConnectedDataStream.java @@ -22,7 +22,7 @@ import java.io.Serializable; import org.apache.commons.lang3.SerializationException; import org.apache.commons.lang3.SerializationUtils; import org.apache.flink.api.common.functions.Function; -import org.apache.flink.api.java.functions.RichFlatMapFunction; +import org.apache.flink.api.common.functions.RichFlatMapFunction; import org.apache.flink.streaming.api.JobGraphBuilder; import org.apache.flink.streaming.api.environment.StreamExecutionEnvironment; import org.apache.flink.streaming.api.function.co.CoFlatMapFunction; diff --git a/flink-addons/flink-streaming/flink-streaming-core/src/main/java/org/apache/flink/streaming/api/datastream/DataStream.java b/flink-addons/flink-streaming/flink-streaming-core/src/main/java/org/apache/flink/streaming/api/datastream/DataStream.java index 0e1ae575612..970415b2f92 100644 --- a/flink-addons/flink-streaming/flink-streaming-core/src/main/java/org/apache/flink/streaming/api/datastream/DataStream.java +++ b/flink-addons/flink-streaming/flink-streaming-core/src/main/java/org/apache/flink/streaming/api/datastream/DataStream.java @@ -28,10 +28,10 @@ import org.apache.flink.api.common.functions.FlatMapFunction; import org.apache.flink.api.common.functions.Function; import org.apache.flink.api.common.functions.MapFunction; import org.apache.flink.api.common.functions.ReduceFunction; -import org.apache.flink.api.java.functions.RichFilterFunction; -import org.apache.flink.api.java.functions.RichFlatMapFunction; -import org.apache.flink.api.java.functions.RichMapFunction; -import org.apache.flink.api.java.functions.RichReduceFunction; +import org.apache.flink.api.common.functions.RichFilterFunction; +import org.apache.flink.api.common.functions.RichFlatMapFunction; +import org.apache.flink.api.common.functions.RichMapFunction; +import org.apache.flink.api.common.functions.RichReduceFunction; import org.apache.flink.api.java.typeutils.TupleTypeInfo; import org.apache.flink.streaming.api.JobGraphBuilder; import org.apache.flink.streaming.api.environment.StreamExecutionEnvironment; diff --git a/flink-addons/flink-streaming/flink-streaming-core/src/main/java/org/apache/flink/streaming/api/datastream/GroupedDataStream.java b/flink-addons/flink-streaming/flink-streaming-core/src/main/java/org/apache/flink/streaming/api/datastream/GroupedDataStream.java index 8978b1991bc..5261222865b 100755 --- a/flink-addons/flink-streaming/flink-streaming-core/src/main/java/org/apache/flink/streaming/api/datastream/GroupedDataStream.java +++ b/flink-addons/flink-streaming/flink-streaming-core/src/main/java/org/apache/flink/streaming/api/datastream/GroupedDataStream.java @@ -18,7 +18,7 @@ package org.apache.flink.streaming.api.datastream; import org.apache.flink.api.common.functions.ReduceFunction; -import org.apache.flink.api.java.functions.RichReduceFunction; +import org.apache.flink.api.common.functions.RichReduceFunction; import org.apache.flink.streaming.api.function.aggregation.AggregationFunction; import org.apache.flink.streaming.api.invokable.operator.GroupedReduceInvokable; import org.apache.flink.streaming.partitioner.StreamPartitioner; diff --git a/flink-addons/flink-streaming/flink-streaming-core/src/test/java/org/apache/flink/streaming/api/IterateTest.java b/flink-addons/flink-streaming/flink-streaming-core/src/test/java/org/apache/flink/streaming/api/IterateTest.java index 83f7e8e0d2b..fc14256255e 100644 --- a/flink-addons/flink-streaming/flink-streaming-core/src/test/java/org/apache/flink/streaming/api/IterateTest.java +++ b/flink-addons/flink-streaming/flink-streaming-core/src/test/java/org/apache/flink/streaming/api/IterateTest.java @@ -19,7 +19,7 @@ package org.apache.flink.streaming.api; import static org.junit.Assert.assertTrue; -import org.apache.flink.api.java.functions.RichFlatMapFunction; +import org.apache.flink.api.common.functions.RichFlatMapFunction; import org.apache.flink.streaming.api.datastream.DataStream; import org.apache.flink.streaming.api.datastream.IterativeDataStream; import org.apache.flink.streaming.api.environment.LocalStreamEnvironment; diff --git a/flink-addons/flink-streaming/flink-streaming-core/src/test/java/org/apache/flink/streaming/api/invokable/operator/GroupedBatchGroupReduceTest.java b/flink-addons/flink-streaming/flink-streaming-core/src/test/java/org/apache/flink/streaming/api/invokable/operator/GroupedBatchGroupReduceTest.java index 096141da4bd..a64089368b2 100755 --- a/flink-addons/flink-streaming/flink-streaming-core/src/test/java/org/apache/flink/streaming/api/invokable/operator/GroupedBatchGroupReduceTest.java +++ b/flink-addons/flink-streaming/flink-streaming-core/src/test/java/org/apache/flink/streaming/api/invokable/operator/GroupedBatchGroupReduceTest.java @@ -23,7 +23,7 @@ import java.util.Arrays; import java.util.List; import org.apache.flink.api.common.functions.GroupReduceFunction; -import org.apache.flink.api.java.functions.RichGroupReduceFunction; +import org.apache.flink.api.common.functions.RichGroupReduceFunction; import org.apache.flink.api.java.tuple.Tuple2; import org.apache.flink.configuration.Configuration; import org.apache.flink.streaming.util.MockInvokable; diff --git a/flink-addons/flink-streaming/flink-streaming-core/src/test/java/org/apache/flink/streaming/api/streamcomponent/StreamComponentTest.java b/flink-addons/flink-streaming/flink-streaming-core/src/test/java/org/apache/flink/streaming/api/streamcomponent/StreamComponentTest.java index bfc8c7a31e9..3e79401e406 100644 --- a/flink-addons/flink-streaming/flink-streaming-core/src/test/java/org/apache/flink/streaming/api/streamcomponent/StreamComponentTest.java +++ b/flink-addons/flink-streaming/flink-streaming-core/src/test/java/org/apache/flink/streaming/api/streamcomponent/StreamComponentTest.java @@ -25,7 +25,7 @@ import java.util.HashMap; import java.util.HashSet; import java.util.Map; -import org.apache.flink.api.java.functions.RichMapFunction; +import org.apache.flink.api.common.functions.RichMapFunction; import org.apache.flink.api.java.tuple.Tuple1; import org.apache.flink.api.java.tuple.Tuple2; import org.apache.flink.streaming.api.datastream.DataStream; diff --git a/flink-addons/flink-streaming/flink-streaming-core/src/test/java/org/apache/flink/streaming/util/serialization/TypeSerializationTest.java b/flink-addons/flink-streaming/flink-streaming-core/src/test/java/org/apache/flink/streaming/util/serialization/TypeSerializationTest.java index c260077be67..cf89e2fc0f9 100644 --- a/flink-addons/flink-streaming/flink-streaming-core/src/test/java/org/apache/flink/streaming/util/serialization/TypeSerializationTest.java +++ b/flink-addons/flink-streaming/flink-streaming-core/src/test/java/org/apache/flink/streaming/util/serialization/TypeSerializationTest.java @@ -21,7 +21,7 @@ import static org.junit.Assert.assertEquals; import static org.junit.Assert.assertNotNull; import org.apache.commons.lang3.SerializationUtils; -import org.apache.flink.api.java.functions.RichMapFunction; +import org.apache.flink.api.common.functions.RichMapFunction; import org.junit.Test; public class TypeSerializationTest { diff --git a/flink-addons/flink-streaming/flink-streaming-examples/src/main/java/org/apache/flink/streaming/examples/ml/IncrementalLearningSkeleton.java b/flink-addons/flink-streaming/flink-streaming-examples/src/main/java/org/apache/flink/streaming/examples/ml/IncrementalLearningSkeleton.java index 3218c471ccf..d3cd2e6c240 100755 --- a/flink-addons/flink-streaming/flink-streaming-examples/src/main/java/org/apache/flink/streaming/examples/ml/IncrementalLearningSkeleton.java +++ b/flink-addons/flink-streaming/flink-streaming-examples/src/main/java/org/apache/flink/streaming/examples/ml/IncrementalLearningSkeleton.java @@ -134,4 +134,7 @@ public class IncrementalLearningSkeleton { env.execute(); } + + + } \ No newline at end of file diff --git a/flink-clients/src/test/java/org/apache/flink/client/testjar/WordCount.java b/flink-clients/src/test/java/org/apache/flink/client/testjar/WordCount.java index 2b64b84f06e..244760227e2 100644 --- a/flink-clients/src/test/java/org/apache/flink/client/testjar/WordCount.java +++ b/flink-clients/src/test/java/org/apache/flink/client/testjar/WordCount.java @@ -19,7 +19,7 @@ package org.apache.flink.client.testjar; import org.apache.flink.api.java.aggregation.Aggregations; -import org.apache.flink.api.java.functions.RichFlatMapFunction; +import org.apache.flink.api.common.functions.RichFlatMapFunction; import org.apache.flink.api.java.tuple.Tuple2; import org.apache.flink.util.Collector; diff --git a/flink-compiler/src/test/java/org/apache/flink/compiler/BranchingPlansCompilerTest.java b/flink-compiler/src/test/java/org/apache/flink/compiler/BranchingPlansCompilerTest.java index fd2f4228996..63ed9073f42 100644 --- a/flink-compiler/src/test/java/org/apache/flink/compiler/BranchingPlansCompilerTest.java +++ b/flink-compiler/src/test/java/org/apache/flink/compiler/BranchingPlansCompilerTest.java @@ -30,7 +30,7 @@ import org.apache.flink.api.common.Plan; import org.apache.flink.api.java.DataSet; import org.apache.flink.api.java.ExecutionEnvironment; import org.apache.flink.api.java.operators.IterativeDataSet; -import org.apache.flink.api.java.functions.RichJoinFunction; +import org.apache.flink.api.common.functions.RichJoinFunction; import org.apache.flink.api.java.record.operators.BulkIteration; import org.apache.flink.api.java.record.operators.CoGroupOperator; import org.apache.flink.api.java.record.operators.CrossOperator; diff --git a/flink-compiler/src/test/java/org/apache/flink/compiler/CachedMatchStrategyCompilerTest.java b/flink-compiler/src/test/java/org/apache/flink/compiler/CachedMatchStrategyCompilerTest.java index 1deead32faf..5c27b999fba 100644 --- a/flink-compiler/src/test/java/org/apache/flink/compiler/CachedMatchStrategyCompilerTest.java +++ b/flink-compiler/src/test/java/org/apache/flink/compiler/CachedMatchStrategyCompilerTest.java @@ -28,7 +28,7 @@ import org.apache.flink.api.common.operators.base.GenericDataSourceBase; import org.apache.flink.api.java.DataSet; import org.apache.flink.api.java.ExecutionEnvironment; import org.apache.flink.api.java.operators.IterativeDataSet; -import org.apache.flink.api.java.functions.RichJoinFunction; +import org.apache.flink.api.common.functions.RichJoinFunction; import org.apache.flink.api.java.tuple.Tuple3; import org.apache.flink.compiler.dag.TempMode; import org.apache.flink.compiler.plan.DualInputPlanNode; diff --git a/flink-compiler/src/test/java/org/apache/flink/compiler/CoGroupSolutionSetFirstTest.java b/flink-compiler/src/test/java/org/apache/flink/compiler/CoGroupSolutionSetFirstTest.java index 54af34497f5..7c3fc276b97 100644 --- a/flink-compiler/src/test/java/org/apache/flink/compiler/CoGroupSolutionSetFirstTest.java +++ b/flink-compiler/src/test/java/org/apache/flink/compiler/CoGroupSolutionSetFirstTest.java @@ -18,8 +18,8 @@ package org.apache.flink.compiler; -import org.apache.flink.api.java.functions.RichCoGroupFunction; -import org.apache.flink.api.java.functions.RichMapFunction; +import org.apache.flink.api.common.functions.RichCoGroupFunction; +import org.apache.flink.api.common.functions.RichMapFunction; import org.junit.Assert; import org.junit.Test; import org.apache.flink.api.common.Plan; diff --git a/flink-compiler/src/test/java/org/apache/flink/compiler/GroupReduceCompilationTest.java b/flink-compiler/src/test/java/org/apache/flink/compiler/GroupReduceCompilationTest.java index 9f636839fcf..0441a6fa873 100644 --- a/flink-compiler/src/test/java/org/apache/flink/compiler/GroupReduceCompilationTest.java +++ b/flink-compiler/src/test/java/org/apache/flink/compiler/GroupReduceCompilationTest.java @@ -20,7 +20,7 @@ package org.apache.flink.compiler; import org.apache.flink.api.common.Plan; import org.apache.flink.api.common.operators.util.FieldList; -import org.apache.flink.api.java.functions.RichGroupReduceFunction; +import org.apache.flink.api.common.functions.RichGroupReduceFunction; import org.apache.flink.api.java.functions.KeySelector; import org.apache.flink.api.java.operators.GroupReduceOperator; import org.apache.flink.api.java.tuple.Tuple2; diff --git a/flink-compiler/src/test/java/org/apache/flink/compiler/IterationsCompilerTest.java b/flink-compiler/src/test/java/org/apache/flink/compiler/IterationsCompilerTest.java index 562336f388d..3d423ef2433 100644 --- a/flink-compiler/src/test/java/org/apache/flink/compiler/IterationsCompilerTest.java +++ b/flink-compiler/src/test/java/org/apache/flink/compiler/IterationsCompilerTest.java @@ -20,7 +20,7 @@ package org.apache.flink.compiler; import static org.junit.Assert.*; -import org.apache.flink.api.java.functions.RichFlatMapFunction; +import org.apache.flink.api.common.functions.RichFlatMapFunction; import org.junit.Test; import org.apache.flink.api.common.Plan; @@ -29,9 +29,9 @@ import org.apache.flink.api.java.operators.DeltaIteration; import org.apache.flink.api.java.ExecutionEnvironment; import org.apache.flink.api.java.operators.IterativeDataSet; import org.apache.flink.api.java.aggregation.Aggregations; -import org.apache.flink.api.java.functions.RichGroupReduceFunction; -import org.apache.flink.api.java.functions.RichJoinFunction; -import org.apache.flink.api.java.functions.RichMapFunction; +import org.apache.flink.api.common.functions.RichGroupReduceFunction; +import org.apache.flink.api.common.functions.RichJoinFunction; +import org.apache.flink.api.common.functions.RichMapFunction; import org.apache.flink.api.java.functions.FunctionAnnotation.ConstantFields; import org.apache.flink.api.java.tuple.Tuple1; import org.apache.flink.api.java.tuple.Tuple2; diff --git a/flink-compiler/src/test/java/org/apache/flink/compiler/ReduceCompilationTest.java b/flink-compiler/src/test/java/org/apache/flink/compiler/ReduceCompilationTest.java index fb8ae8d5fd8..5a69b3c833d 100644 --- a/flink-compiler/src/test/java/org/apache/flink/compiler/ReduceCompilationTest.java +++ b/flink-compiler/src/test/java/org/apache/flink/compiler/ReduceCompilationTest.java @@ -22,7 +22,7 @@ package org.apache.flink.compiler; import org.apache.flink.api.common.Plan; import org.apache.flink.api.common.operators.util.FieldList; import org.apache.flink.api.java.functions.KeySelector; -import org.apache.flink.api.java.functions.RichReduceFunction; +import org.apache.flink.api.common.functions.RichReduceFunction; import org.apache.flink.api.java.tuple.Tuple2; import org.junit.Test; import org.apache.flink.api.java.DataSet; diff --git a/flink-compiler/src/test/java/org/apache/flink/compiler/UnionPropertyPropagationTest.java b/flink-compiler/src/test/java/org/apache/flink/compiler/UnionPropertyPropagationTest.java index 1020c8b894a..f4e5614a75a 100644 --- a/flink-compiler/src/test/java/org/apache/flink/compiler/UnionPropertyPropagationTest.java +++ b/flink-compiler/src/test/java/org/apache/flink/compiler/UnionPropertyPropagationTest.java @@ -25,7 +25,7 @@ import org.apache.flink.api.common.Plan; import org.apache.flink.api.common.operators.base.FlatMapOperatorBase; import org.apache.flink.api.common.operators.base.GroupReduceOperatorBase; import org.apache.flink.api.java.aggregation.Aggregations; -import org.apache.flink.api.java.functions.RichFlatMapFunction; +import org.apache.flink.api.common.functions.RichFlatMapFunction; import org.apache.flink.api.java.record.operators.FileDataSink; import org.apache.flink.api.java.record.operators.FileDataSource; import org.apache.flink.api.java.record.operators.ReduceOperator; diff --git a/flink-compiler/src/test/java/org/apache/flink/compiler/WorksetIterationsJavaApiCompilerTest.java b/flink-compiler/src/test/java/org/apache/flink/compiler/WorksetIterationsJavaApiCompilerTest.java index a7c615224b5..3e73565255f 100644 --- a/flink-compiler/src/test/java/org/apache/flink/compiler/WorksetIterationsJavaApiCompilerTest.java +++ b/flink-compiler/src/test/java/org/apache/flink/compiler/WorksetIterationsJavaApiCompilerTest.java @@ -29,9 +29,9 @@ import org.apache.flink.api.common.operators.util.FieldList; import org.apache.flink.api.java.DataSet; import org.apache.flink.api.java.operators.DeltaIteration; import org.apache.flink.api.java.ExecutionEnvironment; -import org.apache.flink.api.java.functions.RichGroupReduceFunction; -import org.apache.flink.api.java.functions.RichJoinFunction; -import org.apache.flink.api.java.functions.RichMapFunction; +import org.apache.flink.api.common.functions.RichGroupReduceFunction; +import org.apache.flink.api.common.functions.RichJoinFunction; +import org.apache.flink.api.common.functions.RichMapFunction; import org.apache.flink.api.java.tuple.Tuple3; import org.apache.flink.compiler.plan.DualInputPlanNode; import org.apache.flink.compiler.plan.OptimizedPlan; diff --git a/flink-compiler/src/test/java/org/apache/flink/compiler/testfunctions/DummyFlatJoinFunction.java b/flink-compiler/src/test/java/org/apache/flink/compiler/testfunctions/DummyFlatJoinFunction.java index 2388db464c6..1cb3ac0c203 100644 --- a/flink-compiler/src/test/java/org/apache/flink/compiler/testfunctions/DummyFlatJoinFunction.java +++ b/flink-compiler/src/test/java/org/apache/flink/compiler/testfunctions/DummyFlatJoinFunction.java @@ -19,7 +19,7 @@ package org.apache.flink.compiler.testfunctions; -import org.apache.flink.api.java.functions.RichFlatJoinFunction; +import org.apache.flink.api.common.functions.RichFlatJoinFunction; import org.apache.flink.util.Collector; public class DummyFlatJoinFunction extends RichFlatJoinFunction { diff --git a/flink-compiler/src/test/java/org/apache/flink/compiler/testfunctions/IdentityGroupReducer.java b/flink-compiler/src/test/java/org/apache/flink/compiler/testfunctions/IdentityGroupReducer.java index e06e3ba0c1d..4163e33407b 100644 --- a/flink-compiler/src/test/java/org/apache/flink/compiler/testfunctions/IdentityGroupReducer.java +++ b/flink-compiler/src/test/java/org/apache/flink/compiler/testfunctions/IdentityGroupReducer.java @@ -19,8 +19,8 @@ package org.apache.flink.compiler.testfunctions; -import org.apache.flink.api.java.functions.RichGroupReduceFunction; -import org.apache.flink.api.java.functions.RichGroupReduceFunction.Combinable; +import org.apache.flink.api.common.functions.RichGroupReduceFunction; +import org.apache.flink.api.common.functions.RichGroupReduceFunction.Combinable; import org.apache.flink.util.Collector; diff --git a/flink-compiler/src/test/java/org/apache/flink/compiler/testfunctions/IdentityMapper.java b/flink-compiler/src/test/java/org/apache/flink/compiler/testfunctions/IdentityMapper.java index 29fc2c8e231..e797cf6a5a9 100644 --- a/flink-compiler/src/test/java/org/apache/flink/compiler/testfunctions/IdentityMapper.java +++ b/flink-compiler/src/test/java/org/apache/flink/compiler/testfunctions/IdentityMapper.java @@ -19,7 +19,7 @@ package org.apache.flink.compiler.testfunctions; -import org.apache.flink.api.java.functions.RichMapFunction; +import org.apache.flink.api.common.functions.RichMapFunction; public class IdentityMapper extends RichMapFunction { diff --git a/flink-compiler/src/test/java/org/apache/flink/compiler/testfunctions/SelectOneReducer.java b/flink-compiler/src/test/java/org/apache/flink/compiler/testfunctions/SelectOneReducer.java index 7ce267f597d..71764dc6ca8 100644 --- a/flink-compiler/src/test/java/org/apache/flink/compiler/testfunctions/SelectOneReducer.java +++ b/flink-compiler/src/test/java/org/apache/flink/compiler/testfunctions/SelectOneReducer.java @@ -18,7 +18,7 @@ package org.apache.flink.compiler.testfunctions; -import org.apache.flink.api.java.functions.RichReduceFunction; +import org.apache.flink.api.common.functions.RichReduceFunction; public class SelectOneReducer extends RichReduceFunction { diff --git a/flink-compiler/src/test/java/org/apache/flink/compiler/testfunctions/Top1GroupReducer.java b/flink-compiler/src/test/java/org/apache/flink/compiler/testfunctions/Top1GroupReducer.java index b1a0e2d4c00..f1b4e0be518 100644 --- a/flink-compiler/src/test/java/org/apache/flink/compiler/testfunctions/Top1GroupReducer.java +++ b/flink-compiler/src/test/java/org/apache/flink/compiler/testfunctions/Top1GroupReducer.java @@ -18,8 +18,8 @@ package org.apache.flink.compiler.testfunctions; -import org.apache.flink.api.java.functions.RichGroupReduceFunction; -import org.apache.flink.api.java.functions.RichGroupReduceFunction.Combinable; +import org.apache.flink.api.common.functions.RichGroupReduceFunction; +import org.apache.flink.api.common.functions.RichGroupReduceFunction.Combinable; import org.apache.flink.util.Collector; diff --git a/flink-java/src/main/java/org/apache/flink/api/java/functions/RichCoGroupFunction.java b/flink-core/src/main/java/org/apache/flink/api/common/functions/RichCoGroupFunction.java similarity index 97% rename from flink-java/src/main/java/org/apache/flink/api/java/functions/RichCoGroupFunction.java rename to flink-core/src/main/java/org/apache/flink/api/common/functions/RichCoGroupFunction.java index 3169622f88a..b221cb76df5 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/functions/RichCoGroupFunction.java +++ b/flink-core/src/main/java/org/apache/flink/api/common/functions/RichCoGroupFunction.java @@ -16,7 +16,7 @@ * limitations under the License. */ -package org.apache.flink.api.java.functions; +package org.apache.flink.api.common.functions; import org.apache.flink.api.common.functions.AbstractRichFunction; import org.apache.flink.api.common.functions.CoGroupFunction; diff --git a/flink-java/src/main/java/org/apache/flink/api/java/functions/RichCrossFunction.java b/flink-core/src/main/java/org/apache/flink/api/common/functions/RichCrossFunction.java similarity index 97% rename from flink-java/src/main/java/org/apache/flink/api/java/functions/RichCrossFunction.java rename to flink-core/src/main/java/org/apache/flink/api/common/functions/RichCrossFunction.java index 58be279c75c..38ad961cd4f 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/functions/RichCrossFunction.java +++ b/flink-core/src/main/java/org/apache/flink/api/common/functions/RichCrossFunction.java @@ -16,7 +16,7 @@ * limitations under the License. */ -package org.apache.flink.api.java.functions; +package org.apache.flink.api.common.functions; import org.apache.flink.api.common.functions.AbstractRichFunction; import org.apache.flink.api.common.functions.CrossFunction; diff --git a/flink-java/src/main/java/org/apache/flink/api/java/functions/RichFilterFunction.java b/flink-core/src/main/java/org/apache/flink/api/common/functions/RichFilterFunction.java similarity index 97% rename from flink-java/src/main/java/org/apache/flink/api/java/functions/RichFilterFunction.java rename to flink-core/src/main/java/org/apache/flink/api/common/functions/RichFilterFunction.java index 9057a0f195b..b3308665143 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/functions/RichFilterFunction.java +++ b/flink-core/src/main/java/org/apache/flink/api/common/functions/RichFilterFunction.java @@ -16,7 +16,7 @@ * limitations under the License. */ -package org.apache.flink.api.java.functions; +package org.apache.flink.api.common.functions; import org.apache.flink.api.common.functions.AbstractRichFunction; import org.apache.flink.api.common.functions.FilterFunction; diff --git a/flink-java/src/main/java/org/apache/flink/api/java/functions/RichFlatCombineFunction.java b/flink-core/src/main/java/org/apache/flink/api/common/functions/RichFlatCombineFunction.java similarity index 97% rename from flink-java/src/main/java/org/apache/flink/api/java/functions/RichFlatCombineFunction.java rename to flink-core/src/main/java/org/apache/flink/api/common/functions/RichFlatCombineFunction.java index a5d45aa77c9..5d6717d1f53 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/functions/RichFlatCombineFunction.java +++ b/flink-core/src/main/java/org/apache/flink/api/common/functions/RichFlatCombineFunction.java @@ -16,7 +16,7 @@ * limitations under the License. */ -package org.apache.flink.api.java.functions; +package org.apache.flink.api.common.functions; import org.apache.flink.api.common.functions.AbstractRichFunction; diff --git a/flink-java/src/main/java/org/apache/flink/api/java/functions/RichFlatJoinFunction.java b/flink-core/src/main/java/org/apache/flink/api/common/functions/RichFlatJoinFunction.java similarity index 97% rename from flink-java/src/main/java/org/apache/flink/api/java/functions/RichFlatJoinFunction.java rename to flink-core/src/main/java/org/apache/flink/api/common/functions/RichFlatJoinFunction.java index 6918364588d..e1ec0f1586c 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/functions/RichFlatJoinFunction.java +++ b/flink-core/src/main/java/org/apache/flink/api/common/functions/RichFlatJoinFunction.java @@ -16,7 +16,7 @@ * limitations under the License. */ -package org.apache.flink.api.java.functions; +package org.apache.flink.api.common.functions; import org.apache.flink.api.common.functions.AbstractRichFunction; import org.apache.flink.api.common.functions.FlatJoinFunction; diff --git a/flink-java/src/main/java/org/apache/flink/api/java/functions/RichFlatMapFunction.java b/flink-core/src/main/java/org/apache/flink/api/common/functions/RichFlatMapFunction.java similarity index 97% rename from flink-java/src/main/java/org/apache/flink/api/java/functions/RichFlatMapFunction.java rename to flink-core/src/main/java/org/apache/flink/api/common/functions/RichFlatMapFunction.java index c045508d73f..f3ec4a7c5e6 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/functions/RichFlatMapFunction.java +++ b/flink-core/src/main/java/org/apache/flink/api/common/functions/RichFlatMapFunction.java @@ -16,7 +16,7 @@ * limitations under the License. */ -package org.apache.flink.api.java.functions; +package org.apache.flink.api.common.functions; import org.apache.flink.api.common.functions.AbstractRichFunction; import org.apache.flink.api.common.functions.FlatMapFunction; diff --git a/flink-java/src/main/java/org/apache/flink/api/java/functions/RichGroupReduceFunction.java b/flink-core/src/main/java/org/apache/flink/api/common/functions/RichGroupReduceFunction.java similarity index 93% rename from flink-java/src/main/java/org/apache/flink/api/java/functions/RichGroupReduceFunction.java rename to flink-core/src/main/java/org/apache/flink/api/common/functions/RichGroupReduceFunction.java index 9198aeba94e..6c7edff6346 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/functions/RichGroupReduceFunction.java +++ b/flink-core/src/main/java/org/apache/flink/api/common/functions/RichGroupReduceFunction.java @@ -16,17 +16,13 @@ * limitations under the License. */ -package org.apache.flink.api.java.functions; +package org.apache.flink.api.common.functions; import java.lang.annotation.ElementType; import java.lang.annotation.Retention; import java.lang.annotation.RetentionPolicy; import java.lang.annotation.Target; -import org.apache.flink.api.common.functions.AbstractRichFunction; -import org.apache.flink.api.common.functions.FlatCombineFunction; -import org.apache.flink.api.common.functions.GroupReduceFunction; -import org.apache.flink.api.common.functions.RichFunction; import org.apache.flink.util.Collector; /** diff --git a/flink-java/src/main/java/org/apache/flink/api/java/functions/RichJoinFunction.java b/flink-core/src/main/java/org/apache/flink/api/common/functions/RichJoinFunction.java similarity index 97% rename from flink-java/src/main/java/org/apache/flink/api/java/functions/RichJoinFunction.java rename to flink-core/src/main/java/org/apache/flink/api/common/functions/RichJoinFunction.java index a0c28ee5be8..9139fc47965 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/functions/RichJoinFunction.java +++ b/flink-core/src/main/java/org/apache/flink/api/common/functions/RichJoinFunction.java @@ -16,7 +16,7 @@ * limitations under the License. */ -package org.apache.flink.api.java.functions; +package org.apache.flink.api.common.functions; import org.apache.flink.api.common.functions.AbstractRichFunction; import org.apache.flink.api.common.functions.JoinFunction; diff --git a/flink-java/src/main/java/org/apache/flink/api/java/functions/RichMapFunction.java b/flink-core/src/main/java/org/apache/flink/api/common/functions/RichMapFunction.java similarity index 97% rename from flink-java/src/main/java/org/apache/flink/api/java/functions/RichMapFunction.java rename to flink-core/src/main/java/org/apache/flink/api/common/functions/RichMapFunction.java index f6f53567c1e..30b6666c38b 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/functions/RichMapFunction.java +++ b/flink-core/src/main/java/org/apache/flink/api/common/functions/RichMapFunction.java @@ -16,7 +16,7 @@ * limitations under the License. */ -package org.apache.flink.api.java.functions; +package org.apache.flink.api.common.functions; import org.apache.flink.api.common.functions.AbstractRichFunction; import org.apache.flink.api.common.functions.MapFunction; diff --git a/flink-java/src/main/java/org/apache/flink/api/java/functions/RichMapPartitionFunction.java b/flink-core/src/main/java/org/apache/flink/api/common/functions/RichMapPartitionFunction.java similarity index 97% rename from flink-java/src/main/java/org/apache/flink/api/java/functions/RichMapPartitionFunction.java rename to flink-core/src/main/java/org/apache/flink/api/common/functions/RichMapPartitionFunction.java index 4a3beab297f..7fce2a88155 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/functions/RichMapPartitionFunction.java +++ b/flink-core/src/main/java/org/apache/flink/api/common/functions/RichMapPartitionFunction.java @@ -16,7 +16,7 @@ * limitations under the License. */ -package org.apache.flink.api.java.functions; +package org.apache.flink.api.common.functions; import org.apache.flink.api.common.functions.AbstractRichFunction; import org.apache.flink.api.common.functions.MapPartitionFunction; diff --git a/flink-java/src/main/java/org/apache/flink/api/java/functions/RichReduceFunction.java b/flink-core/src/main/java/org/apache/flink/api/common/functions/RichReduceFunction.java similarity index 97% rename from flink-java/src/main/java/org/apache/flink/api/java/functions/RichReduceFunction.java rename to flink-core/src/main/java/org/apache/flink/api/common/functions/RichReduceFunction.java index a63f8dc739e..c63051090fc 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/functions/RichReduceFunction.java +++ b/flink-core/src/main/java/org/apache/flink/api/common/functions/RichReduceFunction.java @@ -16,7 +16,7 @@ * limitations under the License. */ -package org.apache.flink.api.java.functions; +package org.apache.flink.api.common.functions; import org.apache.flink.api.common.functions.AbstractRichFunction; import org.apache.flink.api.common.functions.ReduceFunction; diff --git a/flink-examples/flink-java-examples/src/main/java/org/apache/flink/example/java/clustering/KMeans.java b/flink-examples/flink-java-examples/src/main/java/org/apache/flink/example/java/clustering/KMeans.java index 3c31af6c128..3bea458825e 100644 --- a/flink-examples/flink-java-examples/src/main/java/org/apache/flink/example/java/clustering/KMeans.java +++ b/flink-examples/flink-java-examples/src/main/java/org/apache/flink/example/java/clustering/KMeans.java @@ -23,7 +23,7 @@ import java.util.Collection; import org.apache.flink.api.common.functions.MapFunction; import org.apache.flink.api.common.functions.ReduceFunction; -import org.apache.flink.api.java.functions.RichMapFunction; +import org.apache.flink.api.common.functions.RichMapFunction; import org.apache.flink.api.java.tuple.Tuple2; import org.apache.flink.api.java.tuple.Tuple3; import org.apache.flink.configuration.Configuration; diff --git a/flink-examples/flink-java-examples/src/main/java/org/apache/flink/example/java/ml/LinearRegression.java b/flink-examples/flink-java-examples/src/main/java/org/apache/flink/example/java/ml/LinearRegression.java index d65a809f87b..6ef6270b5b8 100644 --- a/flink-examples/flink-java-examples/src/main/java/org/apache/flink/example/java/ml/LinearRegression.java +++ b/flink-examples/flink-java-examples/src/main/java/org/apache/flink/example/java/ml/LinearRegression.java @@ -24,7 +24,7 @@ import java.util.Collection; import org.apache.flink.api.common.functions.MapFunction; import org.apache.flink.api.common.functions.ReduceFunction; -import org.apache.flink.api.java.functions.RichMapFunction; +import org.apache.flink.api.common.functions.RichMapFunction; import org.apache.flink.api.java.tuple.Tuple2; import org.apache.flink.configuration.Configuration; import org.apache.flink.example.java.ml.util.LinearRegressionData; diff --git a/flink-examples/flink-java-examples/src/main/java/org/apache/flink/example/java/relational/EmptyFieldsCountAccumulator.java b/flink-examples/flink-java-examples/src/main/java/org/apache/flink/example/java/relational/EmptyFieldsCountAccumulator.java index 4bced17c0f6..a379bf876c9 100644 --- a/flink-examples/flink-java-examples/src/main/java/org/apache/flink/example/java/relational/EmptyFieldsCountAccumulator.java +++ b/flink-examples/flink-java-examples/src/main/java/org/apache/flink/example/java/relational/EmptyFieldsCountAccumulator.java @@ -26,7 +26,7 @@ import org.apache.flink.api.common.JobExecutionResult; import org.apache.flink.api.common.accumulators.Accumulator; import org.apache.flink.api.java.DataSet; import org.apache.flink.api.java.ExecutionEnvironment; -import org.apache.flink.api.java.functions.RichFilterFunction; +import org.apache.flink.api.common.functions.RichFilterFunction; import org.apache.flink.api.java.tuple.Tuple; import org.apache.flink.api.java.tuple.Tuple3; import org.apache.flink.configuration.Configuration; diff --git a/flink-java/src/main/java/org/apache/flink/api/java/DataSet.java b/flink-java/src/main/java/org/apache/flink/api/java/DataSet.java index 3dcd7806a1e..4c111ffcafe 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/DataSet.java +++ b/flink-java/src/main/java/org/apache/flink/api/java/DataSet.java @@ -132,13 +132,13 @@ public abstract class DataSet { /** * Applies a Map transformation on a {@link DataSet}.
- * The transformation calls a {@link org.apache.flink.api.java.functions.RichMapFunction} for each element of the DataSet. + * The transformation calls a {@link org.apache.flink.api.common.functions.RichMapFunction} for each element of the DataSet. * Each MapFunction call returns exactly one element. * * @param mapper The MapFunction that is called for each element of the DataSet. * @return A MapOperator that represents the transformed DataSet. * - * @see org.apache.flink.api.java.functions.RichMapFunction + * @see org.apache.flink.api.common.functions.RichMapFunction * @see MapOperator * @see DataSet */ @@ -185,13 +185,13 @@ public abstract class DataSet { /** * Applies a FlatMap transformation on a {@link DataSet}.
- * The transformation calls a {@link org.apache.flink.api.java.functions.RichFlatMapFunction} for each element of the DataSet. + * The transformation calls a {@link org.apache.flink.api.common.functions.RichFlatMapFunction} for each element of the DataSet. * Each FlatMapFunction call can return any number of elements including none. * * @param flatMapper The FlatMapFunction that is called for each element of the DataSet. * @return A FlatMapOperator that represents the transformed DataSet. * - * @see org.apache.flink.api.java.functions.RichFlatMapFunction + * @see org.apache.flink.api.common.functions.RichFlatMapFunction * @see FlatMapOperator * @see DataSet */ @@ -208,14 +208,14 @@ public abstract class DataSet { /** * Applies a Filter transformation on a {@link DataSet}.
- * The transformation calls a {@link org.apache.flink.api.java.functions.RichFilterFunction} for each element of the DataSet + * The transformation calls a {@link org.apache.flink.api.common.functions.RichFilterFunction} for each element of the DataSet * and retains only those element for which the function returns true. Elements for * which the function returns false are filtered. * * @param filter The FilterFunction that is called for each element of the DataSet. * @return A FilterOperator that represents the filtered DataSet. * - * @see org.apache.flink.api.java.functions.RichFilterFunction + * @see org.apache.flink.api.common.functions.RichFilterFunction * @see FilterOperator * @see DataSet */ @@ -311,14 +311,14 @@ public abstract class DataSet { /** * Applies a Reduce transformation on a non-grouped {@link DataSet}.
- * The transformation consecutively calls a {@link org.apache.flink.api.java.functions.RichReduceFunction} + * The transformation consecutively calls a {@link org.apache.flink.api.common.functions.RichReduceFunction} * until only a single element remains which is the result of the transformation. * A ReduceFunction combines two elements into one new element of the same type. * * @param reducer The ReduceFunction that is applied on the DataSet. * @return A ReduceOperator that represents the reduced DataSet. * - * @see org.apache.flink.api.java.functions.RichReduceFunction + * @see org.apache.flink.api.common.functions.RichReduceFunction * @see ReduceOperator * @see DataSet */ @@ -331,14 +331,14 @@ public abstract class DataSet { /** * Applies a GroupReduce transformation on a non-grouped {@link DataSet}.
- * The transformation calls a {@link org.apache.flink.api.java.functions.RichGroupReduceFunction} once with the full DataSet. + * The transformation calls a {@link org.apache.flink.api.common.functions.RichGroupReduceFunction} once with the full DataSet. * The GroupReduceFunction can iterate over all elements of the DataSet and emit any * number of output elements including none. * * @param reducer The GroupReduceFunction that is applied on the DataSet. * @return A GroupReduceOperator that represents the reduced DataSet. * - * @see org.apache.flink.api.java.functions.RichGroupReduceFunction + * @see org.apache.flink.api.common.functions.RichGroupReduceFunction * @see org.apache.flink.api.java.operators.GroupReduceOperator * @see DataSet */ @@ -600,7 +600,7 @@ public abstract class DataSet { * Initiates a CoGroup transformation.
* A CoGroup transformation combines the elements of * two {@link DataSet DataSets} into one DataSet. It groups each DataSet individually on a key and - * gives groups of both DataSets with equal keys together into a {@link org.apache.flink.api.java.functions.RichCoGroupFunction}. + * gives groups of both DataSets with equal keys together into a {@link org.apache.flink.api.common.functions.RichCoGroupFunction}. * If a DataSet has a group with no matching key in the other DataSet, the CoGroupFunction * is called with an empty group for the non-existing group.
* The CoGroupFunction can iterate over the elements of both groups and return any number diff --git a/flink-java/src/main/java/org/apache/flink/api/java/functions/FlatMapIterator.java b/flink-java/src/main/java/org/apache/flink/api/java/functions/FlatMapIterator.java index 0a83235de8e..2030de3436d 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/functions/FlatMapIterator.java +++ b/flink-java/src/main/java/org/apache/flink/api/java/functions/FlatMapIterator.java @@ -20,10 +20,11 @@ package org.apache.flink.api.java.functions; import java.util.Iterator; +import org.apache.flink.api.common.functions.RichFlatMapFunction; import org.apache.flink.util.Collector; /** - * A convenience variant of the {@link RichFlatMapFunction} that returns elements through an iterator, rather then + * A convenience variant of the {@link org.apache.flink.api.common.functions.RichFlatMapFunction} that returns elements through an iterator, rather then * through a collector. In all other respects, it behaves exactly like the FlatMapFunction. *

* The function needs to be serializable, as defined in {@link java.io.Serializable}. diff --git a/flink-java/src/main/java/org/apache/flink/api/java/functions/FunctionAnnotation.java b/flink-java/src/main/java/org/apache/flink/api/java/functions/FunctionAnnotation.java index b94840f4baa..edb1c74ef11 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/functions/FunctionAnnotation.java +++ b/flink-java/src/main/java/org/apache/flink/api/java/functions/FunctionAnnotation.java @@ -64,15 +64,15 @@ import org.apache.flink.api.common.InvalidProgramException; * *

* Be aware that some annotations should only be used for functions with as single input - * ({@link RichMapFunction}, {@link RichReduceFunction}) and some only for stubs with two inputs - * ({@link RichCrossFunction}, {@link RichFlatJoinFunction}, {@link RichCoGroupFunction}). + * ({@link org.apache.flink.api.common.functions.RichMapFunction}, {@link org.apache.flink.api.common.functions.RichReduceFunction}) and some only for stubs with two inputs + * ({@link org.apache.flink.api.common.functions.RichCrossFunction}, {@link org.apache.flink.api.common.functions.RichFlatJoinFunction}, {@link org.apache.flink.api.common.functions.RichCoGroupFunction}). */ public class FunctionAnnotation { /** * This annotation declares that a function leaves certain fields of its input values unmodified and * only "forwards" or "copies" them to the return value. The annotation is applicable to unary - * functions, like for example {@link RichMapFunction}, {@link RichReduceFunction}, or {@link RichFlatMapFunction}. + * functions, like for example {@link org.apache.flink.api.common.functions.RichMapFunction}, {@link org.apache.flink.api.common.functions.RichReduceFunction}, or {@link org.apache.flink.api.common.functions.RichFlatMapFunction}. *

* The following example illustrates a function that keeps the tuple's field zero constant: *

@@ -103,7 +103,7 @@ public class FunctionAnnotation { /** * This annotation declares that a function leaves certain fields of its first input values unmodified and * only "forwards" or "copies" them to the return value. The annotation is applicable to binary - * functions, like for example {@link RichFlatJoinFunction}, {@link RichCoGroupFunction}, or {@link RichCrossFunction}. + * functions, like for example {@link org.apache.flink.api.common.functions.RichFlatJoinFunction}, {@link org.apache.flink.api.common.functions.RichCoGroupFunction}, or {@link org.apache.flink.api.common.functions.RichCrossFunction}. *

* The following example illustrates a join function that copies fields from the first and second input to the * return value: @@ -135,7 +135,7 @@ public class FunctionAnnotation { /** * This annotation declares that a function leaves certain fields of its second input values unmodified and * only "forwards" or "copies" them to the return value. The annotation is applicable to binary - * functions, like for example {@link RichFlatJoinFunction}, {@link RichCoGroupFunction}, or {@link RichCrossFunction}. + * functions, like for example {@link org.apache.flink.api.common.functions.RichFlatJoinFunction}, {@link org.apache.flink.api.common.functions.RichCoGroupFunction}, or {@link org.apache.flink.api.common.functions.RichCrossFunction}. *

* The following example illustrates a join function that copies fields from the first and second input to the * return value: @@ -167,7 +167,7 @@ public class FunctionAnnotation { /** * This annotation declares that a function changes certain fields of its input values, while leaving all * others unmodified and in place in the return value. The annotation is applicable to unary - * functions, like for example {@link RichMapFunction}, {@link RichReduceFunction}, or {@link RichFlatMapFunction}. + * functions, like for example {@link org.apache.flink.api.common.functions.RichMapFunction}, {@link org.apache.flink.api.common.functions.RichReduceFunction}, or {@link org.apache.flink.api.common.functions.RichFlatMapFunction}. *

* The following example illustrates that at the example of a Map function: * @@ -201,7 +201,7 @@ public class FunctionAnnotation { /** * This annotation declares that a function changes certain fields of its first input value, while leaving all * others unmodified and in place in the return value. The annotation is applicable to binary - * functions, like for example {@link RichFlatJoinFunction}, {@link RichCoGroupFunction}, or {@link RichCrossFunction}. + * functions, like for example {@link org.apache.flink.api.common.functions.RichFlatJoinFunction}, {@link org.apache.flink.api.common.functions.RichCoGroupFunction}, or {@link org.apache.flink.api.common.functions.RichCrossFunction}. *

* The following example illustrates a join function that copies fields from the first and second input to the * return value: @@ -238,7 +238,7 @@ public class FunctionAnnotation { /** * This annotation declares that a function changes certain fields of its second input value, while leaving all * others unmodified and in place in the return value. The annotation is applicable to binary - * functions, like for example {@link RichFlatJoinFunction}, {@link RichCoGroupFunction}, or {@link RichCrossFunction}. + * functions, like for example {@link org.apache.flink.api.common.functions.RichFlatJoinFunction}, {@link org.apache.flink.api.common.functions.RichCoGroupFunction}, or {@link org.apache.flink.api.common.functions.RichCrossFunction}. *

* The following example illustrates a join function that copies fields from the first and second input to the * return value: diff --git a/flink-java/src/main/java/org/apache/flink/api/java/functions/GroupReduceIterator.java b/flink-java/src/main/java/org/apache/flink/api/java/functions/GroupReduceIterator.java index 087808d5538..2363e4f8f2a 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/functions/GroupReduceIterator.java +++ b/flink-java/src/main/java/org/apache/flink/api/java/functions/GroupReduceIterator.java @@ -20,6 +20,7 @@ package org.apache.flink.api.java.functions; import java.util.Iterator; +import org.apache.flink.api.common.functions.RichGroupReduceFunction; import org.apache.flink.util.Collector; diff --git a/flink-java/src/main/java/org/apache/flink/api/java/operators/AggregateOperator.java b/flink-java/src/main/java/org/apache/flink/api/java/operators/AggregateOperator.java index d1fed5ec95a..be5fe48a993 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/operators/AggregateOperator.java +++ b/flink-java/src/main/java/org/apache/flink/api/java/operators/AggregateOperator.java @@ -33,8 +33,8 @@ import org.apache.flink.api.java.DataSet; import org.apache.flink.api.java.aggregation.AggregationFunction; import org.apache.flink.api.java.aggregation.AggregationFunctionFactory; import org.apache.flink.api.java.aggregation.Aggregations; -import org.apache.flink.api.java.functions.RichGroupReduceFunction; -import org.apache.flink.api.java.functions.RichGroupReduceFunction.Combinable; +import org.apache.flink.api.common.functions.RichGroupReduceFunction; +import org.apache.flink.api.common.functions.RichGroupReduceFunction.Combinable; import org.apache.flink.api.java.tuple.Tuple; import org.apache.flink.api.java.typeutils.TupleTypeInfo; import org.apache.flink.configuration.Configuration; diff --git a/flink-java/src/main/java/org/apache/flink/api/java/operators/CoGroupOperator.java b/flink-java/src/main/java/org/apache/flink/api/java/operators/CoGroupOperator.java index 81f29e2b48d..ab61d0435d1 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/operators/CoGroupOperator.java +++ b/flink-java/src/main/java/org/apache/flink/api/java/operators/CoGroupOperator.java @@ -452,7 +452,7 @@ public class CoGroupOperator extends TwoInputUdfOperator - * To continue the CoGroup transformation, provide a {@link org.apache.flink.api.java.functions.RichCoGroupFunction} by calling + * To continue the CoGroup transformation, provide a {@link org.apache.flink.api.common.functions.RichCoGroupFunction} by calling * {@link org.apache.flink.api.java.operators.CoGroupOperator.CoGroupOperatorSets.CoGroupOperatorSetsPredicate.CoGroupOperatorWithoutFunction#with(org.apache.flink.api.common.functions.CoGroupFunction)}. * */ @@ -506,13 +506,13 @@ public class CoGroupOperator extends TwoInputUdfOperator + * Finalizes a CoGroup transformation by applying a {@link org.apache.flink.api.common.functions.RichCoGroupFunction} to groups of elements with identical keys.
* Each CoGroupFunction call returns an arbitrary number of keys. * * @param function The CoGroupFunction that is called for all groups of elements with identical keys. * @return An CoGroupOperator that represents the co-grouped result DataSet. * - * @see org.apache.flink.api.java.functions.RichCoGroupFunction + * @see org.apache.flink.api.common.functions.RichCoGroupFunction * @see DataSet */ public CoGroupOperator with(CoGroupFunction function) { diff --git a/flink-java/src/main/java/org/apache/flink/api/java/operators/DistinctOperator.java b/flink-java/src/main/java/org/apache/flink/api/java/operators/DistinctOperator.java index 7d8a28f90d0..7c063cc4f58 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/operators/DistinctOperator.java +++ b/flink-java/src/main/java/org/apache/flink/api/java/operators/DistinctOperator.java @@ -25,8 +25,8 @@ import org.apache.flink.api.common.operators.Operator; import org.apache.flink.api.common.operators.UnaryOperatorInformation; import org.apache.flink.api.common.operators.base.GroupReduceOperatorBase; import org.apache.flink.api.common.operators.base.MapOperatorBase; -import org.apache.flink.api.java.functions.RichGroupReduceFunction; -import org.apache.flink.api.java.functions.RichGroupReduceFunction.Combinable; +import org.apache.flink.api.common.functions.RichGroupReduceFunction; +import org.apache.flink.api.common.functions.RichGroupReduceFunction.Combinable; import org.apache.flink.api.java.operators.translation.KeyExtractingMapper; import org.apache.flink.api.java.operators.translation.PlanUnwrappingReduceGroupOperator; import org.apache.flink.api.java.tuple.Tuple2; diff --git a/flink-java/src/main/java/org/apache/flink/api/java/operators/GroupReduceOperator.java b/flink-java/src/main/java/org/apache/flink/api/java/operators/GroupReduceOperator.java index 9de8bd366b5..a8bdca7a6f1 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/operators/GroupReduceOperator.java +++ b/flink-java/src/main/java/org/apache/flink/api/java/operators/GroupReduceOperator.java @@ -27,7 +27,7 @@ import org.apache.flink.api.common.operators.Ordering; import org.apache.flink.api.common.operators.UnaryOperatorInformation; import org.apache.flink.api.common.operators.base.GroupReduceOperatorBase; import org.apache.flink.api.common.operators.base.MapOperatorBase; -import org.apache.flink.api.java.functions.RichGroupReduceFunction; +import org.apache.flink.api.common.functions.RichGroupReduceFunction; import org.apache.flink.api.java.operators.translation.KeyExtractingMapper; import org.apache.flink.api.java.operators.translation.PlanUnwrappingReduceGroupOperator; import org.apache.flink.api.java.tuple.Tuple2; diff --git a/flink-java/src/main/java/org/apache/flink/api/java/operators/JoinOperator.java b/flink-java/src/main/java/org/apache/flink/api/java/operators/JoinOperator.java index 2875251222c..f242453996c 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/operators/JoinOperator.java +++ b/flink-java/src/main/java/org/apache/flink/api/java/operators/JoinOperator.java @@ -34,7 +34,7 @@ import org.apache.flink.api.common.operators.base.JoinOperatorBase; import org.apache.flink.api.common.operators.base.MapOperatorBase; import org.apache.flink.api.java.DataSet; import org.apache.flink.api.java.operators.DeltaIteration.SolutionSetPlaceHolder; -import org.apache.flink.api.java.functions.RichFlatJoinFunction; +import org.apache.flink.api.common.functions.RichFlatJoinFunction; import org.apache.flink.api.java.functions.KeySelector; import org.apache.flink.api.java.functions.SemanticPropUtil; import org.apache.flink.api.java.functions.UnsupportedLambdaExpressionException; @@ -153,7 +153,7 @@ public abstract class JoinOperator extends TwoInputUdfOperator The type of the second input DataSet of the Join transformation. * @param The type of the result of the Join transformation. * - * @see org.apache.flink.api.java.functions.RichFlatJoinFunction + * @see org.apache.flink.api.common.functions.RichFlatJoinFunction * @see DataSet */ public static class EquiJoin extends JoinOperator { @@ -487,13 +487,13 @@ public abstract class JoinOperator extends TwoInputUdfOperator + * Finalizes a Join transformation by applying a {@link org.apache.flink.api.common.functions.RichFlatJoinFunction} to each pair of joined elements.
* Each JoinFunction call returns exactly one element. * * @param function The JoinFunction that is called for each pair of joined elements. * @return An EquiJoin that represents the joined result DataSet * - * @see org.apache.flink.api.java.functions.RichFlatJoinFunction + * @see org.apache.flink.api.common.functions.RichFlatJoinFunction * @see org.apache.flink.api.java.operators.JoinOperator.EquiJoin * @see DataSet */ diff --git a/flink-java/src/main/java/org/apache/flink/api/java/operators/SingleInputUdfOperator.java b/flink-java/src/main/java/org/apache/flink/api/java/operators/SingleInputUdfOperator.java index dcdbed4c8ab..5f81b62c246 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/operators/SingleInputUdfOperator.java +++ b/flink-java/src/main/java/org/apache/flink/api/java/operators/SingleInputUdfOperator.java @@ -35,8 +35,8 @@ import org.apache.flink.api.java.DataSet; /** * The SingleInputUdfOperator is the base class of all unary operators that execute * user-defined functions (UDFs). The UDFs encapsulated by this operator are naturally UDFs that - * have one input (such as {@link org.apache.flink.api.java.functions.RichMapFunction} or - * {@link org.apache.flink.api.java.functions.RichReduceFunction}). + * have one input (such as {@link org.apache.flink.api.common.functions.RichMapFunction} or + * {@link org.apache.flink.api.common.functions.RichReduceFunction}). *

* This class encapsulates utilities for the UDFs, such as broadcast variables, parameterization * through configuration objects, and semantic properties. diff --git a/flink-java/src/main/java/org/apache/flink/api/java/operators/SortedGrouping.java b/flink-java/src/main/java/org/apache/flink/api/java/operators/SortedGrouping.java index 1647055efe3..ed252c4998c 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/operators/SortedGrouping.java +++ b/flink-java/src/main/java/org/apache/flink/api/java/operators/SortedGrouping.java @@ -70,14 +70,14 @@ public class SortedGrouping extends Grouping { /** * Applies a GroupReduce transformation on a grouped and sorted {@link DataSet}.
- * The transformation calls a {@link org.apache.flink.api.java.functions.RichGroupReduceFunction} for each group of the DataSet. + * The transformation calls a {@link org.apache.flink.api.common.functions.RichGroupReduceFunction} for each group of the DataSet. * A GroupReduceFunction can iterate over all elements of a group and emit any * number of output elements including none. * * @param reducer The GroupReduceFunction that is applied on each group of the DataSet. * @return A GroupReduceOperator that represents the reduced DataSet. * - * @see org.apache.flink.api.java.functions.RichGroupReduceFunction + * @see org.apache.flink.api.common.functions.RichGroupReduceFunction * @see GroupReduceOperator * @see DataSet */ diff --git a/flink-java/src/main/java/org/apache/flink/api/java/operators/TwoInputUdfOperator.java b/flink-java/src/main/java/org/apache/flink/api/java/operators/TwoInputUdfOperator.java index f347fef9f7c..030ad3f4e5e 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/operators/TwoInputUdfOperator.java +++ b/flink-java/src/main/java/org/apache/flink/api/java/operators/TwoInputUdfOperator.java @@ -35,8 +35,8 @@ import org.apache.flink.api.java.DataSet; /** * The TwoInputUdfOperator is the base class of all binary operators that execute * user-defined functions (UDFs). The UDFs encapsulated by this operator are naturally UDFs that - * have two inputs (such as {@link org.apache.flink.api.java.functions.RichJoinFunction} or - * {@link org.apache.flink.api.java.functions.RichCoGroupFunction}). + * have two inputs (such as {@link org.apache.flink.api.common.functions.RichJoinFunction} or + * {@link org.apache.flink.api.common.functions.RichCoGroupFunction}). *

* This class encapsulates utilities for the UDFs, such as broadcast variables, parameterization * through configuration objects, and semantic properties. diff --git a/flink-java/src/main/java/org/apache/flink/api/java/operators/UdfOperator.java b/flink-java/src/main/java/org/apache/flink/api/java/operators/UdfOperator.java index bf33f4e96b9..8af3322345a 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/operators/UdfOperator.java +++ b/flink-java/src/main/java/org/apache/flink/api/java/operators/UdfOperator.java @@ -27,8 +27,8 @@ import org.apache.flink.api.java.DataSet; /** * This interface marks operators as operators that execute user-defined functions (UDFs), such as - * {@link org.apache.flink.api.java.functions.RichMapFunction}, {@link org.apache.flink.api.java.functions.RichReduceFunction}, - * or {@link org.apache.flink.api.java.functions.RichCoGroupFunction}. + * {@link org.apache.flink.api.common.functions.RichMapFunction}, {@link org.apache.flink.api.common.functions.RichReduceFunction}, + * or {@link org.apache.flink.api.common.functions.RichCoGroupFunction}. * The UDF operators stand in contrast to operators that execute built-in operations, like aggregations. */ public interface UdfOperator> { diff --git a/flink-java/src/main/java/org/apache/flink/api/java/operators/UnsortedGrouping.java b/flink-java/src/main/java/org/apache/flink/api/java/operators/UnsortedGrouping.java index fdc86deaaac..54a841bd123 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/operators/UnsortedGrouping.java +++ b/flink-java/src/main/java/org/apache/flink/api/java/operators/UnsortedGrouping.java @@ -97,14 +97,14 @@ public class UnsortedGrouping extends Grouping { /** * Applies a Reduce transformation on a grouped {@link DataSet}.
- * For each group, the transformation consecutively calls a {@link org.apache.flink.api.java.functions.RichReduceFunction} + * For each group, the transformation consecutively calls a {@link org.apache.flink.api.common.functions.RichReduceFunction} * until only a single element for each group remains. * A ReduceFunction combines two elements into one new element of the same type. * * @param reducer The ReduceFunction that is applied on each group of the DataSet. * @return A ReduceOperator that represents the reduced DataSet. * - * @see org.apache.flink.api.java.functions.RichReduceFunction + * @see org.apache.flink.api.common.functions.RichReduceFunction * @see ReduceOperator * @see DataSet */ @@ -117,14 +117,14 @@ public class UnsortedGrouping extends Grouping { /** * Applies a GroupReduce transformation on a grouped {@link DataSet}.
- * The transformation calls a {@link org.apache.flink.api.java.functions.RichGroupReduceFunction} for each group of the DataSet. + * The transformation calls a {@link org.apache.flink.api.common.functions.RichGroupReduceFunction} for each group of the DataSet. * A GroupReduceFunction can iterate over all elements of a group and emit any * number of output elements including none. * * @param reducer The GroupReduceFunction that is applied on each group of the DataSet. * @return A GroupReduceOperator that represents the reduced DataSet. * - * @see org.apache.flink.api.java.functions.RichGroupReduceFunction + * @see org.apache.flink.api.common.functions.RichGroupReduceFunction * @see GroupReduceOperator * @see DataSet */ diff --git a/flink-java/src/main/java/org/apache/flink/api/java/operators/translation/KeyExtractingMapper.java b/flink-java/src/main/java/org/apache/flink/api/java/operators/translation/KeyExtractingMapper.java index c7f65f0f519..68036183d25 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/operators/translation/KeyExtractingMapper.java +++ b/flink-java/src/main/java/org/apache/flink/api/java/operators/translation/KeyExtractingMapper.java @@ -19,7 +19,7 @@ package org.apache.flink.api.java.operators.translation; import org.apache.flink.api.java.functions.KeySelector; -import org.apache.flink.api.java.functions.RichMapFunction; +import org.apache.flink.api.common.functions.RichMapFunction; import org.apache.flink.api.java.tuple.Tuple2; diff --git a/flink-java/src/main/java/org/apache/flink/api/java/operators/translation/KeyRemovingMapper.java b/flink-java/src/main/java/org/apache/flink/api/java/operators/translation/KeyRemovingMapper.java index a6cd8372f6e..8213d1136d3 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/operators/translation/KeyRemovingMapper.java +++ b/flink-java/src/main/java/org/apache/flink/api/java/operators/translation/KeyRemovingMapper.java @@ -18,7 +18,7 @@ package org.apache.flink.api.java.operators.translation; -import org.apache.flink.api.java.functions.RichMapFunction; +import org.apache.flink.api.common.functions.RichMapFunction; import org.apache.flink.api.java.tuple.Tuple2; diff --git a/flink-java/src/main/java/org/apache/flink/api/java/operators/translation/PlanUnwrappingReduceGroupOperator.java b/flink-java/src/main/java/org/apache/flink/api/java/operators/translation/PlanUnwrappingReduceGroupOperator.java index 7e3f0e52522..c222ff27635 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/operators/translation/PlanUnwrappingReduceGroupOperator.java +++ b/flink-java/src/main/java/org/apache/flink/api/java/operators/translation/PlanUnwrappingReduceGroupOperator.java @@ -22,7 +22,7 @@ import org.apache.flink.api.common.functions.FlatCombineFunction; import org.apache.flink.api.common.functions.GroupReduceFunction; import org.apache.flink.api.common.operators.UnaryOperatorInformation; import org.apache.flink.api.common.operators.base.GroupReduceOperatorBase; -import org.apache.flink.api.java.functions.RichGroupReduceFunction; +import org.apache.flink.api.common.functions.RichGroupReduceFunction; import org.apache.flink.api.java.operators.Keys; import org.apache.flink.api.java.tuple.Tuple2; import org.apache.flink.types.TypeInformation; diff --git a/flink-java/src/test/java/org/apache/flink/api/java/functions/SemanticPropertiesTranslationTest.java b/flink-java/src/test/java/org/apache/flink/api/java/functions/SemanticPropertiesTranslationTest.java index 474b0227400..c0f6ba6a4ad 100644 --- a/flink-java/src/test/java/org/apache/flink/api/java/functions/SemanticPropertiesTranslationTest.java +++ b/flink-java/src/test/java/org/apache/flink/api/java/functions/SemanticPropertiesTranslationTest.java @@ -22,6 +22,8 @@ package org.apache.flink.api.java.functions; import static org.junit.Assert.*; import org.apache.flink.api.common.Plan; +import org.apache.flink.api.common.functions.RichJoinFunction; +import org.apache.flink.api.common.functions.RichMapFunction; import org.apache.flink.api.common.operators.DualInputSemanticProperties; import org.apache.flink.api.common.operators.SingleInputSemanticProperties; import org.apache.flink.api.common.operators.base.GenericDataSinkBase; diff --git a/flink-java/src/test/java/org/apache/flink/api/java/operators/translation/DeltaIterationTranslationTest.java b/flink-java/src/test/java/org/apache/flink/api/java/operators/translation/DeltaIterationTranslationTest.java index 71eb6825ab2..e49c54ca01c 100644 --- a/flink-java/src/test/java/org/apache/flink/api/java/operators/translation/DeltaIterationTranslationTest.java +++ b/flink-java/src/test/java/org/apache/flink/api/java/operators/translation/DeltaIterationTranslationTest.java @@ -35,9 +35,9 @@ import org.apache.flink.api.common.operators.base.MapOperatorBase; import org.apache.flink.api.java.DataSet; import org.apache.flink.api.java.operators.DeltaIteration; import org.apache.flink.api.java.ExecutionEnvironment; -import org.apache.flink.api.java.functions.RichCoGroupFunction; -import org.apache.flink.api.java.functions.RichJoinFunction; -import org.apache.flink.api.java.functions.RichMapFunction; +import org.apache.flink.api.common.functions.RichCoGroupFunction; +import org.apache.flink.api.common.functions.RichJoinFunction; +import org.apache.flink.api.common.functions.RichMapFunction; import org.apache.flink.api.java.tuple.Tuple2; import org.apache.flink.api.java.tuple.Tuple3; import org.apache.flink.util.Collector; diff --git a/flink-java/src/test/java/org/apache/flink/api/java/operators/translation/ReduceTranslationTests.java b/flink-java/src/test/java/org/apache/flink/api/java/operators/translation/ReduceTranslationTests.java index 8e457ce6a2e..b770bad0711 100644 --- a/flink-java/src/test/java/org/apache/flink/api/java/operators/translation/ReduceTranslationTests.java +++ b/flink-java/src/test/java/org/apache/flink/api/java/operators/translation/ReduceTranslationTests.java @@ -27,7 +27,7 @@ import org.apache.flink.api.common.operators.base.GenericDataSourceBase; import org.apache.flink.api.common.operators.base.MapOperatorBase; import org.apache.flink.api.common.operators.base.ReduceOperatorBase; import org.apache.flink.api.java.functions.KeySelector; -import org.apache.flink.api.java.functions.RichReduceFunction; +import org.apache.flink.api.common.functions.RichReduceFunction; import org.apache.flink.api.java.tuple.Tuple2; import org.apache.flink.api.java.tuple.Tuple3; import org.apache.flink.api.java.typeutils.TupleTypeInfo; diff --git a/flink-java/src/test/java/org/apache/flink/api/java/type/extractor/TypeExtractorTest.java b/flink-java/src/test/java/org/apache/flink/api/java/type/extractor/TypeExtractorTest.java index 412f7516a18..f21045e7862 100644 --- a/flink-java/src/test/java/org/apache/flink/api/java/type/extractor/TypeExtractorTest.java +++ b/flink-java/src/test/java/org/apache/flink/api/java/type/extractor/TypeExtractorTest.java @@ -24,14 +24,14 @@ import java.io.IOException; import org.apache.flink.api.common.functions.MapFunction; import org.apache.flink.api.common.functions.RuntimeContext; -import org.apache.flink.api.java.functions.RichCoGroupFunction; -import org.apache.flink.api.java.functions.RichCrossFunction; -import org.apache.flink.api.java.functions.RichFlatMapFunction; -import org.apache.flink.api.java.functions.RichGroupReduceFunction; +import org.apache.flink.api.common.functions.RichCoGroupFunction; +import org.apache.flink.api.common.functions.RichCrossFunction; +import org.apache.flink.api.common.functions.RichFlatMapFunction; +import org.apache.flink.api.common.functions.RichGroupReduceFunction; import org.apache.flink.api.java.functions.InvalidTypesException; -import org.apache.flink.api.java.functions.RichFlatJoinFunction; +import org.apache.flink.api.common.functions.RichFlatJoinFunction; import org.apache.flink.api.java.functions.KeySelector; -import org.apache.flink.api.java.functions.RichMapFunction; +import org.apache.flink.api.common.functions.RichMapFunction; import org.apache.flink.api.java.tuple.Tuple; import org.apache.flink.api.java.tuple.Tuple1; import org.apache.flink.api.java.tuple.Tuple2; diff --git a/flink-java8-tests/src/test/java/org/apache/flink/test/javaApiOperators/lambdas/LambdaExtractionTest.java b/flink-java8-tests/src/test/java/org/apache/flink/test/javaApiOperators/lambdas/LambdaExtractionTest.java index 4c8ee23ef6c..fbb7ac85191 100644 --- a/flink-java8-tests/src/test/java/org/apache/flink/test/javaApiOperators/lambdas/LambdaExtractionTest.java +++ b/flink-java8-tests/src/test/java/org/apache/flink/test/javaApiOperators/lambdas/LambdaExtractionTest.java @@ -22,7 +22,7 @@ import static org.junit.Assert.*; import org.apache.flink.api.common.functions.MapFunction; import org.apache.flink.api.common.functions.util.FunctionUtils; -import org.apache.flink.api.java.functions.RichMapFunction; +import org.apache.flink.api.common.functions.RichMapFunction; import org.apache.flink.api.java.tuple.Tuple2; import org.junit.Test; diff --git a/flink-runtime/src/test/java/org/apache/flink/runtime/operators/CachedMatchTaskTest.java b/flink-runtime/src/test/java/org/apache/flink/runtime/operators/CachedMatchTaskTest.java index 3c8bbd044f2..8cbcd58e85a 100644 --- a/flink-runtime/src/test/java/org/apache/flink/runtime/operators/CachedMatchTaskTest.java +++ b/flink-runtime/src/test/java/org/apache/flink/runtime/operators/CachedMatchTaskTest.java @@ -23,12 +23,9 @@ import java.util.List; import java.util.concurrent.atomic.AtomicBoolean; import org.apache.flink.api.common.functions.FlatJoinFunction; -import org.apache.flink.api.java.functions.RichFlatJoinFunction; +import org.apache.flink.api.common.functions.RichFlatJoinFunction; import org.apache.flink.api.java.typeutils.runtime.record.RecordComparator; import org.apache.flink.api.java.typeutils.runtime.record.RecordPairComparatorFactory; -import org.apache.flink.runtime.operators.BuildFirstCachedMatchDriver; -import org.apache.flink.runtime.operators.BuildSecondCachedMatchDriver; -import org.apache.flink.runtime.operators.DriverStrategy; import org.apache.flink.runtime.operators.testutils.DelayingInfinitiveInputIterator; import org.apache.flink.runtime.operators.testutils.DriverTestBase; import org.apache.flink.runtime.operators.testutils.ExpectedTestException; diff --git a/flink-runtime/src/test/java/org/apache/flink/runtime/operators/CoGroupTaskExternalITCase.java b/flink-runtime/src/test/java/org/apache/flink/runtime/operators/CoGroupTaskExternalITCase.java index b65f161f16c..e2cee91433d 100644 --- a/flink-runtime/src/test/java/org/apache/flink/runtime/operators/CoGroupTaskExternalITCase.java +++ b/flink-runtime/src/test/java/org/apache/flink/runtime/operators/CoGroupTaskExternalITCase.java @@ -20,11 +20,9 @@ package org.apache.flink.runtime.operators; import org.junit.Assert; import org.apache.flink.api.common.functions.CoGroupFunction; -import org.apache.flink.api.java.functions.RichCoGroupFunction; +import org.apache.flink.api.common.functions.RichCoGroupFunction; import org.apache.flink.api.java.typeutils.runtime.record.RecordComparator; import org.apache.flink.api.java.typeutils.runtime.record.RecordPairComparatorFactory; -import org.apache.flink.runtime.operators.CoGroupDriver; -import org.apache.flink.runtime.operators.DriverStrategy; import org.apache.flink.runtime.operators.testutils.DriverTestBase; import org.apache.flink.runtime.operators.testutils.UniformRecordGenerator; import org.apache.flink.types.IntValue; diff --git a/flink-runtime/src/test/java/org/apache/flink/runtime/operators/CoGroupTaskTest.java b/flink-runtime/src/test/java/org/apache/flink/runtime/operators/CoGroupTaskTest.java index 968d9479806..e4fed9cbaf8 100644 --- a/flink-runtime/src/test/java/org/apache/flink/runtime/operators/CoGroupTaskTest.java +++ b/flink-runtime/src/test/java/org/apache/flink/runtime/operators/CoGroupTaskTest.java @@ -22,11 +22,9 @@ import java.util.concurrent.atomic.AtomicBoolean; import org.junit.Assert; import org.apache.flink.api.common.functions.CoGroupFunction; -import org.apache.flink.api.java.functions.RichCoGroupFunction; +import org.apache.flink.api.common.functions.RichCoGroupFunction; import org.apache.flink.api.java.typeutils.runtime.record.RecordComparator; import org.apache.flink.api.java.typeutils.runtime.record.RecordPairComparatorFactory; -import org.apache.flink.runtime.operators.CoGroupDriver; -import org.apache.flink.runtime.operators.DriverStrategy; import org.apache.flink.runtime.operators.CoGroupTaskExternalITCase.MockCoGroupStub; import org.apache.flink.runtime.operators.testutils.DelayingInfinitiveInputIterator; import org.apache.flink.runtime.operators.testutils.DriverTestBase; diff --git a/flink-runtime/src/test/java/org/apache/flink/runtime/operators/CombineTaskExternalITCase.java b/flink-runtime/src/test/java/org/apache/flink/runtime/operators/CombineTaskExternalITCase.java index 885a509bce9..e08396e2004 100644 --- a/flink-runtime/src/test/java/org/apache/flink/runtime/operators/CombineTaskExternalITCase.java +++ b/flink-runtime/src/test/java/org/apache/flink/runtime/operators/CombineTaskExternalITCase.java @@ -24,7 +24,7 @@ import java.util.HashMap; import org.junit.Assert; -import org.apache.flink.api.java.functions.RichGroupReduceFunction; +import org.apache.flink.api.common.functions.RichGroupReduceFunction; import org.apache.flink.api.java.typeutils.runtime.record.RecordComparator; import org.apache.flink.runtime.operators.CombineTaskTest.MockCombiningReduceStub; import org.apache.flink.runtime.operators.testutils.DriverTestBase; diff --git a/flink-runtime/src/test/java/org/apache/flink/runtime/operators/CombineTaskTest.java b/flink-runtime/src/test/java/org/apache/flink/runtime/operators/CombineTaskTest.java index 7915d1f7b86..c71e81ca099 100644 --- a/flink-runtime/src/test/java/org/apache/flink/runtime/operators/CombineTaskTest.java +++ b/flink-runtime/src/test/java/org/apache/flink/runtime/operators/CombineTaskTest.java @@ -23,11 +23,9 @@ import java.util.concurrent.atomic.AtomicBoolean; import org.junit.Assert; -import org.apache.flink.api.java.functions.RichGroupReduceFunction; +import org.apache.flink.api.common.functions.RichGroupReduceFunction; import org.apache.flink.api.java.record.operators.ReduceOperator.Combinable; import org.apache.flink.api.java.typeutils.runtime.record.RecordComparator; -import org.apache.flink.runtime.operators.DriverStrategy; -import org.apache.flink.runtime.operators.GroupReduceCombineDriver; import org.apache.flink.runtime.operators.testutils.DelayingInfinitiveInputIterator; import org.apache.flink.runtime.operators.testutils.DiscardingOutputCollector; import org.apache.flink.runtime.operators.testutils.DriverTestBase; diff --git a/flink-runtime/src/test/java/org/apache/flink/runtime/operators/ReduceTaskExternalITCase.java b/flink-runtime/src/test/java/org/apache/flink/runtime/operators/ReduceTaskExternalITCase.java index af58a17aee4..47b0b03868b 100644 --- a/flink-runtime/src/test/java/org/apache/flink/runtime/operators/ReduceTaskExternalITCase.java +++ b/flink-runtime/src/test/java/org/apache/flink/runtime/operators/ReduceTaskExternalITCase.java @@ -25,12 +25,10 @@ import org.junit.Assert; import org.slf4j.Logger; import org.slf4j.LoggerFactory; -import org.apache.flink.api.java.functions.RichGroupReduceFunction; +import org.apache.flink.api.common.functions.RichGroupReduceFunction; import org.apache.flink.api.java.record.operators.ReduceOperator.Combinable; import org.apache.flink.api.java.typeutils.runtime.record.RecordComparator; import org.apache.flink.api.java.typeutils.runtime.record.RecordSerializerFactory; -import org.apache.flink.runtime.operators.DriverStrategy; -import org.apache.flink.runtime.operators.GroupReduceDriver; import org.apache.flink.runtime.operators.sort.CombiningUnilateralSortMerger; import org.apache.flink.runtime.operators.testutils.DriverTestBase; import org.apache.flink.runtime.operators.testutils.UniformRecordGenerator; diff --git a/flink-runtime/src/test/java/org/apache/flink/runtime/operators/ReduceTaskTest.java b/flink-runtime/src/test/java/org/apache/flink/runtime/operators/ReduceTaskTest.java index 97b8950b77a..fb4467179c5 100644 --- a/flink-runtime/src/test/java/org/apache/flink/runtime/operators/ReduceTaskTest.java +++ b/flink-runtime/src/test/java/org/apache/flink/runtime/operators/ReduceTaskTest.java @@ -26,12 +26,10 @@ import org.junit.Assert; import org.slf4j.Logger; import org.slf4j.LoggerFactory; -import org.apache.flink.api.java.functions.RichGroupReduceFunction; +import org.apache.flink.api.common.functions.RichGroupReduceFunction; import org.apache.flink.api.java.record.operators.ReduceOperator.Combinable; import org.apache.flink.api.java.typeutils.runtime.record.RecordComparator; import org.apache.flink.api.java.typeutils.runtime.record.RecordSerializerFactory; -import org.apache.flink.runtime.operators.DriverStrategy; -import org.apache.flink.runtime.operators.GroupReduceDriver; import org.apache.flink.runtime.operators.sort.CombiningUnilateralSortMerger; import org.apache.flink.runtime.operators.testutils.DelayingInfinitiveInputIterator; import org.apache.flink.runtime.operators.testutils.DriverTestBase; diff --git a/flink-runtime/src/test/java/org/apache/flink/runtime/operators/chaining/ChainTaskTest.java b/flink-runtime/src/test/java/org/apache/flink/runtime/operators/chaining/ChainTaskTest.java index 070097ce1d6..e7ca6f37350 100644 --- a/flink-runtime/src/test/java/org/apache/flink/runtime/operators/chaining/ChainTaskTest.java +++ b/flink-runtime/src/test/java/org/apache/flink/runtime/operators/chaining/ChainTaskTest.java @@ -23,7 +23,7 @@ import java.util.List; import org.apache.flink.api.common.functions.GenericCollectorMap; import org.apache.flink.api.common.operators.util.UserCodeClassWrapper; -import org.apache.flink.api.java.functions.RichGroupReduceFunction; +import org.apache.flink.api.common.functions.RichGroupReduceFunction; import org.apache.flink.api.java.typeutils.runtime.record.RecordComparatorFactory; import org.apache.flink.api.java.typeutils.runtime.record.RecordSerializerFactory; import org.apache.flink.configuration.Configuration; diff --git a/flink-runtime/src/test/java/org/apache/flink/runtime/operators/drivers/AllGroupReduceDriverTest.java b/flink-runtime/src/test/java/org/apache/flink/runtime/operators/drivers/AllGroupReduceDriverTest.java index 427dc74833e..e9313b3a651 100644 --- a/flink-runtime/src/test/java/org/apache/flink/runtime/operators/drivers/AllGroupReduceDriverTest.java +++ b/flink-runtime/src/test/java/org/apache/flink/runtime/operators/drivers/AllGroupReduceDriverTest.java @@ -22,7 +22,7 @@ import java.util.Arrays; import java.util.List; import org.apache.flink.api.common.functions.GroupReduceFunction; -import org.apache.flink.api.java.functions.RichGroupReduceFunction; +import org.apache.flink.api.common.functions.RichGroupReduceFunction; import org.apache.flink.api.java.tuple.Tuple2; import org.apache.flink.api.java.typeutils.TypeExtractor; import org.apache.flink.runtime.operators.AllGroupReduceDriver; diff --git a/flink-runtime/src/test/java/org/apache/flink/runtime/operators/drivers/AllReduceDriverTest.java b/flink-runtime/src/test/java/org/apache/flink/runtime/operators/drivers/AllReduceDriverTest.java index b124e511595..0a0f44ec5d3 100644 --- a/flink-runtime/src/test/java/org/apache/flink/runtime/operators/drivers/AllReduceDriverTest.java +++ b/flink-runtime/src/test/java/org/apache/flink/runtime/operators/drivers/AllReduceDriverTest.java @@ -23,7 +23,7 @@ import java.util.Arrays; import java.util.List; import org.apache.flink.api.common.functions.ReduceFunction; -import org.apache.flink.api.java.functions.RichReduceFunction; +import org.apache.flink.api.common.functions.RichReduceFunction; import org.apache.flink.api.java.tuple.Tuple2; import org.apache.flink.api.java.typeutils.TypeExtractor; import org.apache.flink.runtime.operators.AllReduceDriver; diff --git a/flink-runtime/src/test/java/org/apache/flink/runtime/operators/drivers/GroupReduceDriverTest.java b/flink-runtime/src/test/java/org/apache/flink/runtime/operators/drivers/GroupReduceDriverTest.java index a29d4e0f133..4dc64b11951 100644 --- a/flink-runtime/src/test/java/org/apache/flink/runtime/operators/drivers/GroupReduceDriverTest.java +++ b/flink-runtime/src/test/java/org/apache/flink/runtime/operators/drivers/GroupReduceDriverTest.java @@ -22,7 +22,7 @@ import java.util.List; import org.apache.flink.api.common.functions.GroupReduceFunction; import org.apache.flink.api.common.typeutils.TypeComparator; -import org.apache.flink.api.java.functions.RichGroupReduceFunction; +import org.apache.flink.api.common.functions.RichGroupReduceFunction; import org.apache.flink.api.java.tuple.Tuple2; import org.apache.flink.api.java.typeutils.TupleTypeInfo; import org.apache.flink.api.java.typeutils.TypeExtractor; diff --git a/flink-runtime/src/test/java/org/apache/flink/runtime/operators/drivers/ReduceCombineDriverTest.java b/flink-runtime/src/test/java/org/apache/flink/runtime/operators/drivers/ReduceCombineDriverTest.java index ae9c294a930..74b96d8027a 100644 --- a/flink-runtime/src/test/java/org/apache/flink/runtime/operators/drivers/ReduceCombineDriverTest.java +++ b/flink-runtime/src/test/java/org/apache/flink/runtime/operators/drivers/ReduceCombineDriverTest.java @@ -24,7 +24,7 @@ import java.util.List; import org.apache.flink.api.common.functions.ReduceFunction; import org.apache.flink.api.common.typeutils.TypeComparator; -import org.apache.flink.api.java.functions.RichReduceFunction; +import org.apache.flink.api.common.functions.RichReduceFunction; import org.apache.flink.api.java.tuple.Tuple2; import org.apache.flink.api.java.typeutils.TupleTypeInfo; import org.apache.flink.api.java.typeutils.TypeExtractor; diff --git a/flink-runtime/src/test/java/org/apache/flink/runtime/operators/drivers/ReduceDriverTest.java b/flink-runtime/src/test/java/org/apache/flink/runtime/operators/drivers/ReduceDriverTest.java index 28217b4d1df..52ac4b0c9ca 100644 --- a/flink-runtime/src/test/java/org/apache/flink/runtime/operators/drivers/ReduceDriverTest.java +++ b/flink-runtime/src/test/java/org/apache/flink/runtime/operators/drivers/ReduceDriverTest.java @@ -23,7 +23,7 @@ import java.util.List; import org.apache.flink.api.common.functions.ReduceFunction; import org.apache.flink.api.common.typeutils.TypeComparator; -import org.apache.flink.api.java.functions.RichReduceFunction; +import org.apache.flink.api.common.functions.RichReduceFunction; import org.apache.flink.api.java.tuple.Tuple2; import org.apache.flink.api.java.typeutils.TupleTypeInfo; import org.apache.flink.api.java.typeutils.TypeExtractor; diff --git a/flink-runtime/src/test/java/org/apache/flink/runtime/operators/sort/CombiningUnilateralSortMergerITCase.java b/flink-runtime/src/test/java/org/apache/flink/runtime/operators/sort/CombiningUnilateralSortMergerITCase.java index 666ecd56ea8..8d33a98caf6 100644 --- a/flink-runtime/src/test/java/org/apache/flink/runtime/operators/sort/CombiningUnilateralSortMergerITCase.java +++ b/flink-runtime/src/test/java/org/apache/flink/runtime/operators/sort/CombiningUnilateralSortMergerITCase.java @@ -30,7 +30,7 @@ import org.slf4j.LoggerFactory; import org.apache.flink.api.common.typeutils.TypeComparator; import org.apache.flink.api.common.typeutils.TypeSerializer; import org.apache.flink.api.common.typeutils.TypeSerializerFactory; -import org.apache.flink.api.java.functions.RichGroupReduceFunction; +import org.apache.flink.api.common.functions.RichGroupReduceFunction; import org.apache.flink.api.java.typeutils.runtime.record.RecordComparator; import org.apache.flink.api.java.typeutils.runtime.record.RecordSerializerFactory; import org.apache.flink.configuration.Configuration; diff --git a/flink-tests/src/test/java/org/apache/flink/test/compiler/iterations/MultipleJoinsWithSolutionSetCompilerTest.java b/flink-tests/src/test/java/org/apache/flink/test/compiler/iterations/MultipleJoinsWithSolutionSetCompilerTest.java index 1d7cfa635fb..f58e36015d2 100644 --- a/flink-tests/src/test/java/org/apache/flink/test/compiler/iterations/MultipleJoinsWithSolutionSetCompilerTest.java +++ b/flink-tests/src/test/java/org/apache/flink/test/compiler/iterations/MultipleJoinsWithSolutionSetCompilerTest.java @@ -23,9 +23,9 @@ import static org.junit.Assert.fail; import org.apache.flink.api.common.Plan; import org.apache.flink.api.java.aggregation.Aggregations; -import org.apache.flink.api.java.functions.RichFlatMapFunction; -import org.apache.flink.api.java.functions.RichJoinFunction; -import org.apache.flink.api.java.functions.RichMapFunction; +import org.apache.flink.api.common.functions.RichFlatMapFunction; +import org.apache.flink.api.common.functions.RichJoinFunction; +import org.apache.flink.api.common.functions.RichMapFunction; import org.apache.flink.api.java.tuple.Tuple2; import org.apache.flink.api.java.tuple.Tuple3; import org.apache.flink.runtime.operators.DriverStrategy; diff --git a/flink-tests/src/test/java/org/apache/flink/test/iterative/BulkIterationWithAllReducerITCase.java b/flink-tests/src/test/java/org/apache/flink/test/iterative/BulkIterationWithAllReducerITCase.java index 1d0b765b71a..21798bbe0b8 100644 --- a/flink-tests/src/test/java/org/apache/flink/test/iterative/BulkIterationWithAllReducerITCase.java +++ b/flink-tests/src/test/java/org/apache/flink/test/iterative/BulkIterationWithAllReducerITCase.java @@ -22,7 +22,7 @@ import java.util.ArrayList; import java.util.Collection; import java.util.List; -import org.apache.flink.api.java.functions.RichGroupReduceFunction; +import org.apache.flink.api.common.functions.RichGroupReduceFunction; import org.apache.flink.configuration.Configuration; import org.apache.flink.test.util.JavaProgramTestBase; import org.apache.flink.util.Collector; diff --git a/flink-tests/src/test/java/org/apache/flink/test/iterative/CoGroupConnectedComponentsSecondITCase.java b/flink-tests/src/test/java/org/apache/flink/test/iterative/CoGroupConnectedComponentsSecondITCase.java index e4faa77e80e..06a8eafd25d 100644 --- a/flink-tests/src/test/java/org/apache/flink/test/iterative/CoGroupConnectedComponentsSecondITCase.java +++ b/flink-tests/src/test/java/org/apache/flink/test/iterative/CoGroupConnectedComponentsSecondITCase.java @@ -21,9 +21,9 @@ package org.apache.flink.test.iterative; import java.util.ArrayList; import java.util.List; -import org.apache.flink.api.java.functions.RichCoGroupFunction; -import org.apache.flink.api.java.functions.RichFlatMapFunction; -import org.apache.flink.api.java.functions.RichMapFunction; +import org.apache.flink.api.common.functions.RichCoGroupFunction; +import org.apache.flink.api.common.functions.RichFlatMapFunction; +import org.apache.flink.api.common.functions.RichMapFunction; import org.apache.flink.api.java.functions.FunctionAnnotation.ConstantFieldsFirst; import org.apache.flink.api.java.functions.FunctionAnnotation.ConstantFieldsSecond; import org.apache.flink.api.java.tuple.Tuple2; diff --git a/flink-tests/src/test/java/org/apache/flink/test/iterative/DependencyConnectedComponentsITCase.java b/flink-tests/src/test/java/org/apache/flink/test/iterative/DependencyConnectedComponentsITCase.java index a0d26db73c4..2d53f7b6198 100644 --- a/flink-tests/src/test/java/org/apache/flink/test/iterative/DependencyConnectedComponentsITCase.java +++ b/flink-tests/src/test/java/org/apache/flink/test/iterative/DependencyConnectedComponentsITCase.java @@ -21,9 +21,9 @@ package org.apache.flink.test.iterative; import java.util.ArrayList; import java.util.List; -import org.apache.flink.api.java.functions.RichFlatMapFunction; -import org.apache.flink.api.java.functions.RichGroupReduceFunction; -import org.apache.flink.api.java.functions.RichJoinFunction; +import org.apache.flink.api.common.functions.RichFlatMapFunction; +import org.apache.flink.api.common.functions.RichGroupReduceFunction; +import org.apache.flink.api.common.functions.RichJoinFunction; import org.apache.flink.api.java.functions.KeySelector; import org.apache.flink.api.java.tuple.Tuple2; import org.apache.flink.test.util.JavaProgramTestBase; diff --git a/flink-tests/src/test/java/org/apache/flink/test/iterative/aggregators/AggregatorsITCase.java b/flink-tests/src/test/java/org/apache/flink/test/iterative/aggregators/AggregatorsITCase.java index 12eb87bc8c0..98fa54ad64f 100644 --- a/flink-tests/src/test/java/org/apache/flink/test/iterative/aggregators/AggregatorsITCase.java +++ b/flink-tests/src/test/java/org/apache/flink/test/iterative/aggregators/AggregatorsITCase.java @@ -28,8 +28,8 @@ import org.junit.Assert; import org.apache.flink.api.common.aggregators.ConvergenceCriterion; import org.apache.flink.api.common.aggregators.LongSumAggregator; -import org.apache.flink.api.java.functions.RichFlatMapFunction; -import org.apache.flink.api.java.functions.RichMapFunction; +import org.apache.flink.api.common.functions.RichFlatMapFunction; +import org.apache.flink.api.common.functions.RichMapFunction; import org.apache.flink.api.java.tuple.Tuple2; import org.apache.flink.configuration.Configuration; import org.apache.flink.test.javaApiOperators.util.CollectionDataSets; diff --git a/flink-tests/src/test/java/org/apache/flink/test/iterative/aggregators/ConnectedComponentsWithParametrizableAggregatorITCase.java b/flink-tests/src/test/java/org/apache/flink/test/iterative/aggregators/ConnectedComponentsWithParametrizableAggregatorITCase.java index a9f01fb4323..13490401dad 100644 --- a/flink-tests/src/test/java/org/apache/flink/test/iterative/aggregators/ConnectedComponentsWithParametrizableAggregatorITCase.java +++ b/flink-tests/src/test/java/org/apache/flink/test/iterative/aggregators/ConnectedComponentsWithParametrizableAggregatorITCase.java @@ -23,8 +23,8 @@ import java.util.List; import org.apache.flink.api.common.aggregators.LongSumAggregator; import org.apache.flink.api.common.functions.JoinFunction; -import org.apache.flink.api.java.functions.RichFlatMapFunction; -import org.apache.flink.api.java.functions.RichGroupReduceFunction; +import org.apache.flink.api.common.functions.RichFlatMapFunction; +import org.apache.flink.api.common.functions.RichGroupReduceFunction; import org.apache.flink.api.java.tuple.Tuple2; import org.apache.flink.configuration.Configuration; import org.apache.flink.test.util.JavaProgramTestBase; diff --git a/flink-tests/src/test/java/org/apache/flink/test/iterative/aggregators/ConnectedComponentsWithParametrizableConvergenceITCase.java b/flink-tests/src/test/java/org/apache/flink/test/iterative/aggregators/ConnectedComponentsWithParametrizableConvergenceITCase.java index 5e6be50c228..b3559180edb 100644 --- a/flink-tests/src/test/java/org/apache/flink/test/iterative/aggregators/ConnectedComponentsWithParametrizableConvergenceITCase.java +++ b/flink-tests/src/test/java/org/apache/flink/test/iterative/aggregators/ConnectedComponentsWithParametrizableConvergenceITCase.java @@ -23,9 +23,9 @@ import java.util.List; import org.apache.flink.api.common.aggregators.ConvergenceCriterion; import org.apache.flink.api.common.aggregators.LongSumAggregator; -import org.apache.flink.api.java.functions.RichFlatMapFunction; -import org.apache.flink.api.java.functions.RichGroupReduceFunction; -import org.apache.flink.api.java.functions.RichJoinFunction; +import org.apache.flink.api.common.functions.RichFlatMapFunction; +import org.apache.flink.api.common.functions.RichGroupReduceFunction; +import org.apache.flink.api.common.functions.RichJoinFunction; import org.apache.flink.api.java.tuple.Tuple2; import org.apache.flink.configuration.Configuration; import org.apache.flink.test.util.JavaProgramTestBase; diff --git a/flink-tests/src/test/java/org/apache/flink/test/javaApiOperators/CoGroupITCase.java b/flink-tests/src/test/java/org/apache/flink/test/javaApiOperators/CoGroupITCase.java index f0229cb9a68..21897d8ce20 100644 --- a/flink-tests/src/test/java/org/apache/flink/test/javaApiOperators/CoGroupITCase.java +++ b/flink-tests/src/test/java/org/apache/flink/test/javaApiOperators/CoGroupITCase.java @@ -29,7 +29,7 @@ import org.apache.flink.api.common.functions.CoGroupFunction; import org.apache.flink.api.java.DataSet; import org.apache.flink.api.java.ExecutionEnvironment; import org.apache.flink.api.java.functions.KeySelector; -import org.apache.flink.api.java.functions.RichCoGroupFunction; +import org.apache.flink.api.common.functions.RichCoGroupFunction; import org.apache.flink.api.java.tuple.Tuple2; import org.apache.flink.api.java.tuple.Tuple3; import org.apache.flink.api.java.tuple.Tuple5; diff --git a/flink-tests/src/test/java/org/apache/flink/test/javaApiOperators/CrossITCase.java b/flink-tests/src/test/java/org/apache/flink/test/javaApiOperators/CrossITCase.java index 7c264b60aae..215ef2ed61f 100644 --- a/flink-tests/src/test/java/org/apache/flink/test/javaApiOperators/CrossITCase.java +++ b/flink-tests/src/test/java/org/apache/flink/test/javaApiOperators/CrossITCase.java @@ -24,7 +24,7 @@ import java.util.Collection; import java.util.LinkedList; import org.apache.flink.api.common.functions.CrossFunction; -import org.apache.flink.api.java.functions.RichCrossFunction; +import org.apache.flink.api.common.functions.RichCrossFunction; import org.apache.flink.api.java.tuple.Tuple2; import org.apache.flink.api.java.tuple.Tuple3; import org.apache.flink.api.java.tuple.Tuple5; diff --git a/flink-tests/src/test/java/org/apache/flink/test/javaApiOperators/DistinctITCase.java b/flink-tests/src/test/java/org/apache/flink/test/javaApiOperators/DistinctITCase.java index 6e5cd9b6bdb..f28254d3f15 100644 --- a/flink-tests/src/test/java/org/apache/flink/test/javaApiOperators/DistinctITCase.java +++ b/flink-tests/src/test/java/org/apache/flink/test/javaApiOperators/DistinctITCase.java @@ -25,7 +25,7 @@ import java.util.LinkedList; import org.apache.flink.api.java.DataSet; import org.apache.flink.api.java.ExecutionEnvironment; import org.apache.flink.api.java.functions.KeySelector; -import org.apache.flink.api.java.functions.RichMapFunction; +import org.apache.flink.api.common.functions.RichMapFunction; import org.apache.flink.api.java.tuple.Tuple1; import org.apache.flink.api.java.tuple.Tuple2; import org.apache.flink.api.java.tuple.Tuple3; diff --git a/flink-tests/src/test/java/org/apache/flink/test/javaApiOperators/FilterITCase.java b/flink-tests/src/test/java/org/apache/flink/test/javaApiOperators/FilterITCase.java index 6613bc1a63e..d4850b27efa 100644 --- a/flink-tests/src/test/java/org/apache/flink/test/javaApiOperators/FilterITCase.java +++ b/flink-tests/src/test/java/org/apache/flink/test/javaApiOperators/FilterITCase.java @@ -24,7 +24,7 @@ import java.util.Collection; import java.util.LinkedList; import org.apache.flink.api.common.functions.FilterFunction; -import org.apache.flink.api.java.functions.RichFilterFunction; +import org.apache.flink.api.common.functions.RichFilterFunction; import org.apache.flink.api.java.tuple.Tuple3; import org.apache.flink.configuration.Configuration; import org.apache.flink.test.javaApiOperators.util.CollectionDataSets; diff --git a/flink-tests/src/test/java/org/apache/flink/test/javaApiOperators/FlatMapITCase.java b/flink-tests/src/test/java/org/apache/flink/test/javaApiOperators/FlatMapITCase.java index a6dd37722c4..60a8df8ddda 100644 --- a/flink-tests/src/test/java/org/apache/flink/test/javaApiOperators/FlatMapITCase.java +++ b/flink-tests/src/test/java/org/apache/flink/test/javaApiOperators/FlatMapITCase.java @@ -24,7 +24,7 @@ import java.util.Collection; import java.util.LinkedList; import org.apache.flink.api.common.functions.FlatMapFunction; -import org.apache.flink.api.java.functions.RichFlatMapFunction; +import org.apache.flink.api.common.functions.RichFlatMapFunction; import org.apache.flink.api.java.tuple.Tuple3; import org.apache.flink.configuration.Configuration; import org.apache.flink.test.javaApiOperators.util.CollectionDataSets; diff --git a/flink-tests/src/test/java/org/apache/flink/test/javaApiOperators/GroupReduceITCase.java b/flink-tests/src/test/java/org/apache/flink/test/javaApiOperators/GroupReduceITCase.java index 2e00d322cf0..78956909e21 100644 --- a/flink-tests/src/test/java/org/apache/flink/test/javaApiOperators/GroupReduceITCase.java +++ b/flink-tests/src/test/java/org/apache/flink/test/javaApiOperators/GroupReduceITCase.java @@ -27,8 +27,8 @@ import java.util.LinkedList; import org.apache.flink.api.common.functions.GroupReduceFunction; import org.apache.flink.api.common.operators.Order; import org.apache.flink.api.java.functions.KeySelector; -import org.apache.flink.api.java.functions.RichGroupReduceFunction; -import org.apache.flink.api.java.functions.RichMapFunction; +import org.apache.flink.api.common.functions.RichGroupReduceFunction; +import org.apache.flink.api.common.functions.RichMapFunction; import org.apache.flink.api.java.tuple.Tuple2; import org.apache.flink.api.java.tuple.Tuple3; import org.apache.flink.api.java.tuple.Tuple5; diff --git a/flink-tests/src/test/java/org/apache/flink/test/javaApiOperators/JoinITCase.java b/flink-tests/src/test/java/org/apache/flink/test/javaApiOperators/JoinITCase.java index 290b1c234ce..fbac13a286a 100644 --- a/flink-tests/src/test/java/org/apache/flink/test/javaApiOperators/JoinITCase.java +++ b/flink-tests/src/test/java/org/apache/flink/test/javaApiOperators/JoinITCase.java @@ -26,7 +26,7 @@ import java.util.LinkedList; import org.apache.flink.api.common.functions.FlatJoinFunction; import org.apache.flink.api.common.functions.JoinFunction; import org.apache.flink.api.java.functions.KeySelector; -import org.apache.flink.api.java.functions.RichFlatJoinFunction; +import org.apache.flink.api.common.functions.RichFlatJoinFunction; import org.apache.flink.api.java.tuple.Tuple2; import org.apache.flink.api.java.tuple.Tuple3; import org.apache.flink.api.java.tuple.Tuple5; diff --git a/flink-tests/src/test/java/org/apache/flink/test/javaApiOperators/MapITCase.java b/flink-tests/src/test/java/org/apache/flink/test/javaApiOperators/MapITCase.java index 4f1fb1ab74b..75f9721fdee 100644 --- a/flink-tests/src/test/java/org/apache/flink/test/javaApiOperators/MapITCase.java +++ b/flink-tests/src/test/java/org/apache/flink/test/javaApiOperators/MapITCase.java @@ -26,7 +26,7 @@ import java.util.LinkedList; import org.junit.Assert; import org.apache.flink.api.common.functions.MapFunction; -import org.apache.flink.api.java.functions.RichMapFunction; +import org.apache.flink.api.common.functions.RichMapFunction; import org.apache.flink.api.java.tuple.Tuple3; import org.apache.flink.configuration.Configuration; import org.apache.flink.test.javaApiOperators.util.CollectionDataSets; diff --git a/flink-tests/src/test/java/org/apache/flink/test/javaApiOperators/ReduceITCase.java b/flink-tests/src/test/java/org/apache/flink/test/javaApiOperators/ReduceITCase.java index fd7fc9f5a85..f9279bc678e 100644 --- a/flink-tests/src/test/java/org/apache/flink/test/javaApiOperators/ReduceITCase.java +++ b/flink-tests/src/test/java/org/apache/flink/test/javaApiOperators/ReduceITCase.java @@ -27,7 +27,7 @@ import org.apache.flink.api.common.functions.ReduceFunction; import org.apache.flink.api.java.DataSet; import org.apache.flink.api.java.ExecutionEnvironment; import org.apache.flink.api.java.functions.KeySelector; -import org.apache.flink.api.java.functions.RichReduceFunction; +import org.apache.flink.api.common.functions.RichReduceFunction; import org.apache.flink.api.java.tuple.Tuple2; import org.apache.flink.api.java.tuple.Tuple3; import org.apache.flink.api.java.tuple.Tuple5; diff --git a/flink-tests/src/test/java/org/apache/flink/test/javaApiOperators/UnionITCase.java b/flink-tests/src/test/java/org/apache/flink/test/javaApiOperators/UnionITCase.java index a636ba45574..420fb9d03d1 100644 --- a/flink-tests/src/test/java/org/apache/flink/test/javaApiOperators/UnionITCase.java +++ b/flink-tests/src/test/java/org/apache/flink/test/javaApiOperators/UnionITCase.java @@ -23,7 +23,7 @@ import java.io.IOException; import java.util.Collection; import java.util.LinkedList; -import org.apache.flink.api.java.functions.RichFilterFunction; +import org.apache.flink.api.common.functions.RichFilterFunction; import org.apache.flink.api.java.tuple.Tuple3; import org.apache.flink.configuration.Configuration; import org.apache.flink.test.javaApiOperators.util.CollectionDataSets; diff --git a/flink-tests/src/test/java/org/apache/flink/test/util/testjar/KMeansForTest.java b/flink-tests/src/test/java/org/apache/flink/test/util/testjar/KMeansForTest.java index 5390574d964..dbd2f06b827 100644 --- a/flink-tests/src/test/java/org/apache/flink/test/util/testjar/KMeansForTest.java +++ b/flink-tests/src/test/java/org/apache/flink/test/util/testjar/KMeansForTest.java @@ -25,8 +25,8 @@ import java.util.Collection; import org.apache.flink.api.common.Plan; import org.apache.flink.api.common.Program; import org.apache.flink.api.java.functions.KeySelector; -import org.apache.flink.api.java.functions.RichMapFunction; -import org.apache.flink.api.java.functions.RichReduceFunction; +import org.apache.flink.api.common.functions.RichMapFunction; +import org.apache.flink.api.common.functions.RichReduceFunction; import org.apache.flink.api.java.tuple.Tuple2; import org.apache.flink.api.java.tuple.Tuple3; import org.apache.flink.configuration.Configuration; -- GitLab