提交 b9f42e91 编写于 作者: S Stephan Ewen 提交者: Robert Metzger

[FLINK-3995] [build] flink-test-utils also contains the streaming test utilities.

Test utilities include the StreamingMultipleProgramsTestBase and StreamingTestEnvironment.

This moves the ITCases for streaming into 'flink-tests' to achieve that.

This closes #2092
上级 4b71e0e7
......@@ -24,10 +24,11 @@ import org.apache.flink.api.common.typeinfo.TypeInformation;
import org.apache.flink.runtime.client.JobCancellationException;
import org.apache.flink.streaming.api.datastream.DataStream;
import org.apache.flink.streaming.api.environment.StreamExecutionEnvironment;
import org.apache.flink.streaming.api.functions.sink.DiscardingSink;
import org.apache.flink.streaming.connectors.kafka.internals.ZookeeperOffsetHandler;
import org.apache.flink.streaming.connectors.kafka.testutils.DiscardingSink;
import org.apache.flink.streaming.connectors.kafka.testutils.JobManagerCommunicationUtils;
import org.apache.flink.streaming.util.serialization.SimpleStringSchema;
import org.junit.Assert;
import org.junit.Test;
......
......@@ -78,7 +78,7 @@ under the License.
<dependency>
<groupId>org.apache.flink</groupId>
<artifactId>flink-test-utils_2.10</artifactId>
<artifactId>flink-test-utils-junit</artifactId>
<version>${project.version}</version>
<scope>test</scope>
</dependency>
......@@ -95,18 +95,6 @@ under the License.
<build>
<plugins>
<plugin>
<groupId>org.apache.maven.plugins</groupId>
<artifactId>maven-jar-plugin</artifactId>
<executions>
<execution>
<goals>
<goal>test-jar</goal>
</goals>
</execution>
</executions>
</plugin>
<!-- disable fork reuse for the streaming project, because of
incorrect declaration of tests -->
<plugin>
......
......@@ -39,6 +39,7 @@ import org.apache.flink.streaming.api.datastream.SplitStream;
import org.apache.flink.streaming.api.environment.StreamExecutionEnvironment;
import org.apache.flink.streaming.api.functions.co.CoFlatMapFunction;
import org.apache.flink.streaming.api.functions.co.CoMapFunction;
import org.apache.flink.streaming.api.functions.sink.DiscardingSink;
import org.apache.flink.streaming.api.functions.sink.SinkFunction;
import org.apache.flink.streaming.api.functions.windowing.AllWindowFunction;
import org.apache.flink.streaming.api.graph.StreamEdge;
......@@ -57,15 +58,14 @@ import org.apache.flink.streaming.runtime.partitioner.GlobalPartitioner;
import org.apache.flink.streaming.runtime.partitioner.RebalancePartitioner;
import org.apache.flink.streaming.runtime.partitioner.ShufflePartitioner;
import org.apache.flink.streaming.runtime.partitioner.StreamPartitioner;
import org.apache.flink.streaming.util.NoOpSink;
import org.apache.flink.streaming.util.StreamingMultipleProgramsTestBase;
import org.apache.flink.util.Collector;
import org.junit.Test;
import static org.junit.Assert.*;
@SuppressWarnings("serial")
public class DataStreamTest extends StreamingMultipleProgramsTestBase {
public class DataStreamTest {
/**
* Tests union functionality. This ensures that self-unions and unions of streams
......@@ -452,7 +452,7 @@ public class DataStreamTest extends StreamingMultipleProgramsTestBase {
}
});
windowed.addSink(new NoOpSink<Long>());
windowed.addSink(new DiscardingSink<Long>());
DataStreamSink<Long> sink = map.addSink(new SinkFunction<Long>() {
private static final long serialVersionUID = 1L;
......@@ -486,7 +486,7 @@ public class DataStreamTest extends StreamingMultipleProgramsTestBase {
}
DataStreamSource<Long> parallelSource = env.generateSequence(0, 0);
parallelSource.addSink(new NoOpSink<Long>());
parallelSource.addSink(new DiscardingSink<Long>());
assertEquals(7, env.getStreamGraph().getStreamNode(parallelSource.getId()).getParallelism());
parallelSource.setParallelism(3);
......@@ -557,7 +557,7 @@ public class DataStreamTest extends StreamingMultipleProgramsTestBase {
}
};
DataStream<Integer> map = src.map(mapFunction);
map.addSink(new NoOpSink<Integer>());
map.addSink(new DiscardingSink<Integer>());
assertEquals(mapFunction, getFunctionForDataStream(map));
......@@ -569,7 +569,7 @@ public class DataStreamTest extends StreamingMultipleProgramsTestBase {
}
};
DataStream<Integer> flatMap = src.flatMap(flatMapFunction);
flatMap.addSink(new NoOpSink<Integer>());
flatMap.addSink(new DiscardingSink<Integer>());
assertEquals(flatMapFunction, getFunctionForDataStream(flatMap));
FilterFunction<Integer> filterFunction = new FilterFunction<Integer>() {
......@@ -582,7 +582,7 @@ public class DataStreamTest extends StreamingMultipleProgramsTestBase {
DataStream<Integer> unionFilter = map.union(flatMap)
.filter(filterFunction);
unionFilter.addSink(new NoOpSink<Integer>());
unionFilter.addSink(new DiscardingSink<Integer>());
assertEquals(filterFunction, getFunctionForDataStream(unionFilter));
......@@ -606,7 +606,7 @@ public class DataStreamTest extends StreamingMultipleProgramsTestBase {
};
SplitStream<Integer> split = unionFilter.split(outputSelector);
split.select("dummy").addSink(new NoOpSink<Integer>());
split.select("dummy").addSink(new DiscardingSink<Integer>());
List<OutputSelector<?>> outputSelectors = env.getStreamGraph().getStreamNode(unionFilter.getId()).getOutputSelectors();
assertEquals(1, outputSelectors.size());
assertEquals(outputSelector, outputSelectors.get(0));
......@@ -632,7 +632,7 @@ public class DataStreamTest extends StreamingMultipleProgramsTestBase {
}
};
DataStream<String> coMap = connect.map(coMapper);
coMap.addSink(new NoOpSink<String>());
coMap.addSink(new DiscardingSink<String>());
assertEquals(coMapper, getFunctionForDataStream(coMap));
try {
......@@ -772,7 +772,7 @@ public class DataStreamTest extends StreamingMultipleProgramsTestBase {
return null;
}
});
coMap.addSink(new NoOpSink());
coMap.addSink(new DiscardingSink());
return coMap.getId();
}
......
......@@ -31,18 +31,18 @@ import org.apache.flink.api.common.typeinfo.TypeInformation;
import org.apache.flink.streaming.api.datastream.DataStream;
import org.apache.flink.streaming.api.datastream.DataStreamSource;
import org.apache.flink.streaming.api.environment.StreamExecutionEnvironment;
import org.apache.flink.streaming.api.functions.sink.DiscardingSink;
import org.apache.flink.streaming.api.functions.source.FromElementsFunction;
import org.apache.flink.streaming.api.functions.source.SourceFunction;
import org.apache.flink.streaming.api.functions.source.StatefulSequenceSource;
import org.apache.flink.streaming.api.graph.StreamGraph;
import org.apache.flink.streaming.api.operators.AbstractUdfStreamOperator;
import org.apache.flink.streaming.api.operators.StreamOperator;
import org.apache.flink.streaming.util.NoOpSink;
import org.apache.flink.streaming.util.StreamingMultipleProgramsTestBase;
import org.apache.flink.util.SplittableIterator;
import org.junit.Test;
public class StreamExecutionEnvironmentTest extends StreamingMultipleProgramsTestBase {
public class StreamExecutionEnvironmentTest {
@Test
public void fromElementsWithBaseTypeTest1() {
......@@ -73,18 +73,18 @@ public class StreamExecutionEnvironmentTest extends StreamingMultipleProgramsTes
// expected
}
dataStream1.addSink(new NoOpSink<Integer>());
dataStream1.addSink(new DiscardingSink<Integer>());
DataStreamSource<Integer> dataStream2 = env.fromParallelCollection(new DummySplittableIterator<Integer>(),
typeInfo).setParallelism(4);
dataStream2.addSink(new NoOpSink<Integer>());
dataStream2.addSink(new DiscardingSink<Integer>());
String plan = env.getExecutionPlan();
env.getExecutionPlan();
assertEquals("Parallelism of collection source must be 1.", 1, env.getStreamGraph().getStreamNode(dataStream1.getId()).getParallelism());
assertEquals("Parallelism of parallel collection source must be 4.",
4,
4,
env.getStreamGraph().getStreamNode(dataStream2.getId()).getParallelism());
}
catch (Exception e) {
......@@ -109,7 +109,7 @@ public class StreamExecutionEnvironmentTest extends StreamingMultipleProgramsTes
}
};
DataStreamSource<Integer> src1 = env.addSource(srcFun);
src1.addSink(new NoOpSink<Integer>());
src1.addSink(new DiscardingSink<Integer>());
assertEquals(srcFun, getFunctionFromDataSource(src1));
List<Long> list = Arrays.asList(0L, 1L, 2L);
......@@ -135,8 +135,9 @@ public class StreamExecutionEnvironmentTest extends StreamingMultipleProgramsTes
return streamGraph.getStreamNode(dataStream.getId()).getOperator();
}
@SuppressWarnings("unchecked")
private static <T> SourceFunction<T> getFunctionFromDataSource(DataStreamSource<T> dataStreamSource) {
dataStreamSource.addSink(new NoOpSink<T>());
dataStreamSource.addSink(new DiscardingSink<T>());
AbstractUdfStreamOperator<?, ?> operator =
(AbstractUdfStreamOperator<?, ?>) getOperatorFromDataStream(dataStreamSource);
return (SourceFunction<T>) operator.getUserFunction();
......
......@@ -31,12 +31,12 @@ import org.apache.flink.streaming.api.environment.StreamExecutionEnvironment;
import org.apache.flink.streaming.api.functions.co.CoFlatMapFunction;
import org.apache.flink.streaming.api.functions.co.CoMapFunction;
import org.apache.flink.streaming.api.functions.source.SourceFunction;
import org.apache.flink.streaming.util.StreamingMultipleProgramsTestBase;
import org.apache.flink.util.Collector;
import org.junit.Test;
@SuppressWarnings("serial")
public class TypeFillTest extends StreamingMultipleProgramsTestBase {
public class TypeFillTest {
@Test
public void test() {
......
......@@ -25,6 +25,7 @@ import java.util.List;
import org.apache.flink.api.java.tuple.Tuple1;
import org.apache.flink.streaming.api.collector.selector.OutputSelector;
import org.junit.Test;
public class OutputSelectorTest {
......
......@@ -25,6 +25,7 @@ import org.apache.flink.streaming.api.datastream.ConnectedStreams;
import org.apache.flink.streaming.api.datastream.DataStream;
import org.apache.flink.streaming.api.datastream.SingleOutputStreamOperator;
import org.apache.flink.streaming.api.environment.StreamExecutionEnvironment;
import org.apache.flink.streaming.api.functions.sink.DiscardingSink;
import org.apache.flink.streaming.api.operators.AbstractStreamOperator;
import org.apache.flink.streaming.api.operators.OneInputStreamOperator;
import org.apache.flink.streaming.api.operators.Output;
......@@ -39,7 +40,6 @@ import org.apache.flink.streaming.runtime.streamrecord.StreamRecord;
import org.apache.flink.streaming.runtime.tasks.StreamTask;
import org.apache.flink.streaming.util.EvenOddOutputSelector;
import org.apache.flink.streaming.util.NoOpIntMap;
import org.apache.flink.streaming.util.NoOpSink;
import org.junit.Test;
......@@ -49,8 +49,7 @@ import static org.junit.Assert.assertTrue;
/**
* Tests for {@link StreamGraphGenerator}. This only tests correct translation of split/select,
* union, partitioning since the other translation routines are tested already in operation
* specific tests, for example in {@link org.apache.flink.streaming.api.IterateTest} for
* iterations.
* specific tests.
*/
public class StreamGraphGeneratorTest {
......@@ -77,7 +76,7 @@ public class StreamGraphGeneratorTest {
.broadcast()
.map(new NoOpIntMap());
broadcastMap.addSink(new NoOpSink<Integer>());
broadcastMap.addSink(new DiscardingSink<Integer>());
// verify that partitioning is preserved across union and split/select
EvenOddOutputSelector selector1 = new EvenOddOutputSelector();
......@@ -113,7 +112,7 @@ public class StreamGraphGeneratorTest {
SingleOutputStreamOperator<Integer> unionedMap = map1.union(map2).union(map3)
.map(new NoOpIntMap());
unionedMap.addSink(new NoOpSink<Integer>());
unionedMap.addSink(new DiscardingSink<Integer>());
StreamGraph graph = env.getStreamGraph();
......@@ -169,7 +168,7 @@ public class StreamGraphGeneratorTest {
.select("foo")
.map(new NoOpIntMap());
unionedMap.addSink(new NoOpSink<Integer>());
unionedMap.addSink(new DiscardingSink<Integer>());
StreamGraph graph = env.getStreamGraph();
......@@ -207,9 +206,9 @@ public class StreamGraphGeneratorTest {
BasicTypeInfo.INT_TYPE_INFO,
outputTypeConfigurableOperation);
result.addSink(new NoOpSink<Integer>());
result.addSink(new DiscardingSink<Integer>());
StreamGraph graph = env.getStreamGraph();
env.getStreamGraph();
assertEquals(BasicTypeInfo.INT_TYPE_INFO, outputTypeConfigurableOperation.getTypeInformation());
}
......@@ -230,9 +229,9 @@ public class StreamGraphGeneratorTest {
BasicTypeInfo.INT_TYPE_INFO,
outputTypeConfigurableOperation);
result.addSink(new NoOpSink<Integer>());
result.addSink(new DiscardingSink<Integer>());
StreamGraph graph = env.getStreamGraph();
env.getStreamGraph();
assertEquals(BasicTypeInfo.INT_TYPE_INFO, outputTypeConfigurableOperation.getTypeInformation());
}
......
......@@ -17,29 +17,19 @@
package org.apache.flink.streaming.api.operators;
import static org.junit.Assert.assertEquals;
import static org.junit.Assert.fail;
import java.util.HashSet;
import java.util.concurrent.ConcurrentLinkedQueue;
import org.apache.flink.api.common.ExecutionConfig;
import org.apache.flink.api.common.functions.MapFunction;
import org.apache.flink.api.common.typeinfo.TypeInformation;
import org.apache.flink.api.java.tuple.Tuple;
import org.apache.flink.api.java.tuple.Tuple2;
import org.apache.flink.api.java.tuple.Tuple3;
import org.apache.flink.api.java.tuple.Tuple5;
import org.apache.flink.api.java.typeutils.TupleTypeInfo;
import org.apache.flink.api.java.typeutils.TypeExtractor;
import org.apache.flink.api.java.typeutils.runtime.TupleSerializer;
import org.apache.flink.streaming.api.datastream.StreamProjection;
import org.apache.flink.streaming.api.environment.StreamExecutionEnvironment;
import org.apache.flink.streaming.api.functions.sink.SinkFunction;
import org.apache.flink.streaming.api.watermark.Watermark;
import org.apache.flink.streaming.runtime.streamrecord.StreamRecord;
import org.apache.flink.streaming.util.OneInputStreamOperatorTestHarness;
import org.apache.flink.streaming.util.StreamingMultipleProgramsTestBase;
import org.apache.flink.streaming.util.TestHarnessUtil;
import org.junit.Test;
......@@ -52,7 +42,7 @@ import org.junit.Test;
* <li>Watermarks are correctly forwarded</li>
* </ul>
*/
public class StreamProjectTest extends StreamingMultipleProgramsTestBase {
public class StreamProjectTest {
@Test
public void testProject() throws Exception {
......@@ -91,47 +81,4 @@ public class StreamProjectTest extends StreamingMultipleProgramsTestBase {
TestHarnessUtil.assertOutputEquals("Output was not correct.", expectedOutput, testHarness.getOutput());
}
// tests using projection from the API without explicitly specifying the types
private static HashSet<Tuple2<Long, Double>> expected = new HashSet<Tuple2<Long, Double>>();
private static HashSet<Tuple2<Long, Double>> actual = new HashSet<Tuple2<Long, Double>>();
@Test
public void APIWithoutTypesTest() {
for (Long i = 1L; i < 11L; i++) {
expected.add(new Tuple2<Long, Double>(i, i.doubleValue()));
}
StreamExecutionEnvironment env = StreamExecutionEnvironment.getExecutionEnvironment();
env.setParallelism(1);
env.generateSequence(1, 10).map(new MapFunction<Long, Tuple3<Long, Character, Double>>() {
private static final long serialVersionUID = 1L;
@Override
public Tuple3<Long, Character, Double> map(Long value) throws Exception {
return new Tuple3<Long, Character, Double>(value, 'c', value.doubleValue());
}
})
.project(0, 2)
.addSink(new SinkFunction<Tuple>() {
private static final long serialVersionUID = 1L;
@Override
@SuppressWarnings("unchecked")
public void invoke(Tuple value) throws Exception {
actual.add( (Tuple2<Long,Double>) value);
}
});
try {
env.execute();
} catch (Exception e) {
fail(e.getMessage());
}
assertEquals(expected, actual);
}
}
///*
// * Licensed to the Apache Software Foundation (ASF) under one or more
// * contributor license agreements. See the NOTICE file distributed with
// * this work for additional information regarding copyright ownership.
// * The ASF licenses this file to You under the Apache License, Version 2.0
// * (the "License"); you may not use this file except in compliance with
// * the License. You may obtain a copy of the License at
// *
// * http://www.apache.org/licenses/LICENSE-2.0
// *
// * Unless required by applicable law or agreed to in writing, software
// * distributed under the License is distributed on an "AS IS" BASIS,
// * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
// * See the License for the specific language governing permissions and
// * limitations under the License.
// */
//
//package org.apache.flink.streaming.api.operators.co;
//
//import static org.junit.Assert.assertEquals;
//
//import java.util.Arrays;
//import java.util.List;
//
//import org.apache.flink.api.java.functions.KeySelector;
//import org.apache.flink.api.java.tuple.Tuple2;
//import org.apache.flink.api.java.tuple.Tuple3;
//import org.apache.flink.streaming.api.functions.co.CoReduceFunction;
//import org.apache.flink.streaming.api.operators.co.CoStreamGroupedReduce;
//import org.apache.flink.streaming.util.MockCoContext;
//import org.junit.Test;
//
//public class CoGroupedReduceTest {
//
// private final static class MyCoReduceFunction implements
// CoReduceFunction<Tuple3<String, String, String>, Tuple2<Integer, Integer>, String> {
// private static final long serialVersionUID = 1L;
//
// @Override
// public Tuple3<String, String, String> reduce1(Tuple3<String, String, String> value1,
// Tuple3<String, String, String> value2) {
// return new Tuple3<String, String, String>(value1.f0, value1.f1 + value2.f1, value1.f2);
// }
//
// @Override
// public Tuple2<Integer, Integer> reduce2(Tuple2<Integer, Integer> value1,
// Tuple2<Integer, Integer> value2) {
// return new Tuple2<Integer, Integer>(value1.f0, value1.f1 + value2.f1);
// }
//
// @Override
// public String map1(Tuple3<String, String, String> value) {
// return value.f1;
// }
//
// @Override
// public String map2(Tuple2<Integer, Integer> value) {
// return value.f1.toString();
// }
// }
//
// @SuppressWarnings("unchecked")
// @Test
// public void coGroupedReduceTest() {
// Tuple3<String, String, String> word1 = new Tuple3<String, String, String>("a", "word1", "b");
// Tuple3<String, String, String> word2 = new Tuple3<String, String, String>("b", "word2", "a");
// Tuple3<String, String, String> word3 = new Tuple3<String, String, String>("a", "word3", "a");
// Tuple2<Integer, Integer> int1 = new Tuple2<Integer, Integer>(2, 1);
// Tuple2<Integer, Integer> int2 = new Tuple2<Integer, Integer>(1, 2);
// Tuple2<Integer, Integer> int3 = new Tuple2<Integer, Integer>(0, 3);
// Tuple2<Integer, Integer> int4 = new Tuple2<Integer, Integer>(2, 4);
// Tuple2<Integer, Integer> int5 = new Tuple2<Integer, Integer>(1, 5);
//
// KeySelector<Tuple3<String, String, String>, ?> keySelector0 = new KeySelector<Tuple3<String, String, String>, String>() {
//
// private static final long serialVersionUID = 1L;
//
// @Override
// public String getKey(Tuple3<String, String, String> value) throws Exception {
// return value.f0;
// }
// };
//
// KeySelector<Tuple2<Integer, Integer>, ?> keySelector1 = new KeySelector<Tuple2<Integer, Integer>, Integer>() {
//
// private static final long serialVersionUID = 1L;
//
// @Override
// public Integer getKey(Tuple2<Integer, Integer> value) throws Exception {
// return value.f0;
// }
// };
//
// KeySelector<Tuple3<String, String, String>, ?> keySelector2 = new KeySelector<Tuple3<String, String, String>, String>() {
//
// private static final long serialVersionUID = 1L;
//
// @Override
// public String getKey(Tuple3<String, String, String> value) throws Exception {
// return value.f2;
// }
// };
//
// CoStreamGroupedReduce<Tuple3<String, String, String>, Tuple2<Integer, Integer>, String> invokable = new CoStreamGroupedReduce<Tuple3<String, String, String>, Tuple2<Integer, Integer>, String>(
// new MyCoReduceFunction(), keySelector0, keySelector1);
//
// List<String> expected = Arrays.asList("word1", "1", "word2", "2", "word1word3", "3", "5",
// "7");
//
// List<String> actualList = MockCoContext.createAndExecute(invokable,
// Arrays.asList(word1, word2, word3), Arrays.asList(int1, int2, int3, int4, int5));
//
// assertEquals(expected, actualList);
//
// invokable = new CoStreamGroupedReduce<Tuple3<String, String, String>, Tuple2<Integer, Integer>, String>(
// new MyCoReduceFunction(), keySelector2, keySelector1);
//
// expected = Arrays.asList("word1", "1", "word2", "2", "word2word3", "3", "5", "7");
//
// actualList = MockCoContext.createAndExecute(invokable, Arrays.asList(word1, word2, word3),
// Arrays.asList(int1, int2, int3, int4, int5));
//
// assertEquals(expected, actualList);
// }
//}
///*
// * Licensed to the Apache Software Foundation (ASF) under one or more
// * contributor license agreements. See the NOTICE file distributed with
// * this work for additional information regarding copyright ownership.
// * The ASF licenses this file to You under the Apache License, Version 2.0
// * (the "License"); you may not use this file except in compliance with
// * the License. You may obtain a copy of the License at
// *
// * http://www.apache.org/licenses/LICENSE-2.0
// *
// * Unless required by applicable law or agreed to in writing, software
// * distributed under the License is distributed on an "AS IS" BASIS,
// * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
// * See the License for the specific language governing permissions and
// * limitations under the License.
// */
//
//package org.apache.flink.streaming.api.operators.co;
//
//import static org.junit.Assert.assertEquals;
//
//import java.util.ArrayList;
//import java.util.HashSet;
//import java.util.List;
//import java.util.Set;
//
//import org.apache.flink.api.java.tuple.Tuple2;
//import org.apache.flink.streaming.api.functions.co.CoWindowFunction;
//import org.apache.flink.streaming.api.operators.co.CoStreamWindow;
//import org.apache.flink.streaming.api.windowing.helper.Timestamp;
//import org.apache.flink.streaming.api.windowing.helper.TimestampWrapper;
//import org.apache.flink.streaming.util.MockCoContext;
//import org.apache.flink.util.Collector;
//import org.junit.Test;
//
//public class CoWindowTest {
//
// public static final class MyCoGroup1 implements CoWindowFunction<Integer, Integer, Integer> {
//
// private static final long serialVersionUID = 1L;
//
// @SuppressWarnings("unused")
// @Override
// public void coWindow(List<Integer> first, List<Integer> second, Collector<Integer> out)
// throws Exception {
// Integer count1 = 0;
// for (Integer i : first) {
// count1++;
// }
// Integer count2 = 0;
// for (Integer i : second) {
// count2++;
// }
// out.collect(count1);
// out.collect(count2);
//
// }
//
// }
//
// public static final class MyCoGroup2 implements
// CoWindowFunction<Tuple2<Integer, Integer>, Tuple2<Integer, Integer>, Integer> {
//
// private static final long serialVersionUID = 1L;
//
// @Override
// public void coWindow(List<Tuple2<Integer, Integer>> first,
// List<Tuple2<Integer, Integer>> second, Collector<Integer> out) throws Exception {
//
// Set<Integer> firstElements = new HashSet<Integer>();
// for (Tuple2<Integer, Integer> value : first) {
// firstElements.add(value.f1);
// }
// for (Tuple2<Integer, Integer> value : second) {
// if (firstElements.contains(value.f1)) {
// out.collect(value.f1);
// }
// }
//
// }
//
// }
//
// private static final class MyTS1 implements Timestamp<Integer> {
//
// private static final long serialVersionUID = 1L;
//
// @Override
// public long getTimestamp(Integer value) {
// return value;
// }
//
// }
//
// private static final class MyTS2 implements Timestamp<Tuple2<Integer, Integer>> {
//
// private static final long serialVersionUID = 1L;
//
// @Override
// public long getTimestamp(Tuple2<Integer, Integer> value) {
// return value.f0;
// }
//
// }
//
// @Test
// public void coWindowGroupReduceTest2() throws Exception {
//
// CoStreamWindow<Integer, Integer, Integer> invokable1 = new CoStreamWindow<Integer, Integer, Integer>(
// new MyCoGroup1(), 2, 1, new TimestampWrapper<Integer>(new MyTS1(), 1),
// new TimestampWrapper<Integer>(new MyTS1(), 1));
//
// // Windowsize 2, slide 1
// // 1,2|2,3|3,4|4,5
//
// List<Integer> input11 = new ArrayList<Integer>();
// input11.add(1);
// input11.add(1);
// input11.add(2);
// input11.add(3);
// input11.add(3);
//
// List<Integer> input12 = new ArrayList<Integer>();
// input12.add(1);
// input12.add(2);
// input12.add(3);
// input12.add(3);
// input12.add(5);
//
// // Windows: (1,1,2)(1,1,2)|(2,3,3)(2,3,3)|(3,3)(3,3)|(5)(5)
// // expected output: 3,2|3,3|2,2|0,1
//
// List<Integer> expected1 = new ArrayList<Integer>();
// expected1.add(3);
// expected1.add(2);
// expected1.add(3);
// expected1.add(3);
// expected1.add(2);
// expected1.add(2);
// expected1.add(0);
// expected1.add(1);
//
// List<Integer> actual1 = MockCoContext.createAndExecute(invokable1, input11, input12);
// assertEquals(expected1, actual1);
//
// CoStreamWindow<Tuple2<Integer, Integer>, Tuple2<Integer, Integer>, Integer> invokable2 = new CoStreamWindow<Tuple2<Integer, Integer>, Tuple2<Integer, Integer>, Integer>(
// new MyCoGroup2(), 2, 3, new TimestampWrapper<Tuple2<Integer, Integer>>(new MyTS2(),
// 1), new TimestampWrapper<Tuple2<Integer, Integer>>(new MyTS2(), 1));
//
// // WindowSize 2, slide 3
// // 1,2|4,5|7,8|
//
// List<Tuple2<Integer, Integer>> input21 = new ArrayList<Tuple2<Integer, Integer>>();
// input21.add(new Tuple2<Integer, Integer>(1, 1));
// input21.add(new Tuple2<Integer, Integer>(1, 2));
// input21.add(new Tuple2<Integer, Integer>(2, 3));
// input21.add(new Tuple2<Integer, Integer>(3, 4));
// input21.add(new Tuple2<Integer, Integer>(3, 5));
// input21.add(new Tuple2<Integer, Integer>(4, 6));
// input21.add(new Tuple2<Integer, Integer>(4, 7));
// input21.add(new Tuple2<Integer, Integer>(5, 8));
//
// List<Tuple2<Integer, Integer>> input22 = new ArrayList<Tuple2<Integer, Integer>>();
// input22.add(new Tuple2<Integer, Integer>(1, 1));
// input22.add(new Tuple2<Integer, Integer>(2, 0));
// input22.add(new Tuple2<Integer, Integer>(2, 2));
// input22.add(new Tuple2<Integer, Integer>(3, 9));
// input22.add(new Tuple2<Integer, Integer>(3, 4));
// input22.add(new Tuple2<Integer, Integer>(4, 10));
// input22.add(new Tuple2<Integer, Integer>(5, 8));
// input22.add(new Tuple2<Integer, Integer>(5, 7));
//
// List<Integer> expected2 = new ArrayList<Integer>();
// expected2.add(1);
// expected2.add(2);
// expected2.add(8);
// expected2.add(7);
//
// List<Integer> actual2 = MockCoContext.createAndExecute(invokable2, input21, input22);
// assertEquals(expected2, actual2);
// }
//}
/*
* Licensed to the Apache Software Foundation (ASF) under one or more
* contributor license agreements. See the NOTICE file distributed with
* this work for additional information regarding copyright ownership.
* The ASF licenses this file to You under the Apache License, Version 2.0
* (the "License"); you may not use this file except in compliance with
* the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.flink.streaming.api.outputformat;
import org.apache.flink.api.common.functions.MapFunction;
import org.apache.flink.api.java.tuple.Tuple2;
import org.apache.flink.streaming.api.datastream.DataStream;
import org.apache.flink.streaming.api.environment.StreamExecutionEnvironment;
import org.apache.flink.streaming.util.SocketOutputTestBase;
import org.apache.flink.streaming.util.serialization.SimpleStringSchema;
import org.apache.flink.test.testdata.WordCountData;
import org.junit.Ignore;
@Ignore
//This test sometimes failes most likely due to the behaviour
//of the socket. Disabled for now.
public class SocketOutputFormatITCase extends SocketOutputTestBase {
@Override
protected void testProgram() throws Exception {
StreamExecutionEnvironment env = StreamExecutionEnvironment.getExecutionEnvironment();
DataStream<String> text = env.fromElements(WordCountData.TEXT);
DataStream<String> counts =
text.flatMap(new CsvOutputFormatITCase.Tokenizer())
.keyBy(0).sum(1).map(new MapFunction<Tuple2<String, Integer>, String>() {
@Override
public String map(Tuple2<String, Integer> value) throws Exception {
return value.toString() + "\n";
}
});
counts.writeToSocket(HOST, port, new SimpleStringSchema());
env.execute("WriteToSocketTest");
}
}
/*
* Licensed to the Apache Software Foundation (ASF) under one or more
* contributor license agreements. See the NOTICE file distributed with
* this work for additional information regarding copyright ownership.
* The ASF licenses this file to You under the Apache License, Version 2.0
* (the "License"); you may not use this file except in compliance with
* the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.flink.streaming.api.streamtask;
import static org.junit.Assert.assertEquals;
import static org.junit.Assert.fail;
import java.util.Arrays;
import java.util.Collections;
import java.util.HashMap;
import java.util.HashSet;
import java.util.Map;
import java.util.Set;
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;
import org.apache.flink.streaming.api.environment.LocalStreamEnvironment;
import org.apache.flink.streaming.api.environment.StreamExecutionEnvironment;
import org.apache.flink.streaming.api.functions.co.CoMapFunction;
import org.apache.flink.streaming.api.functions.sink.SinkFunction;
import org.apache.flink.streaming.api.functions.source.SourceFunction;
import org.apache.flink.streaming.util.StreamingMultipleProgramsTestBase;
import org.apache.flink.streaming.util.TestStreamEnvironment;
import org.junit.Test;
public class StreamVertexTest extends StreamingMultipleProgramsTestBase {
private static Map<Integer, Integer> data = new HashMap<Integer, Integer>();
public static class MySource implements SourceFunction<Tuple1<Integer>> {
private static final long serialVersionUID = 1L;
private Tuple1<Integer> tuple = new Tuple1<Integer>(0);
private int i = 0;
@Override
public void run(SourceContext<Tuple1<Integer>> ctx) throws Exception {
for (int i = 0; i < 10; i++) {
tuple.f0 = i;
ctx.collect(tuple);
}
}
@Override
public void cancel() {
}
}
public static class MyTask extends RichMapFunction<Tuple1<Integer>, Tuple2<Integer, Integer>> {
private static final long serialVersionUID = 1L;
@Override
public Tuple2<Integer, Integer> map(Tuple1<Integer> value) throws Exception {
Integer i = value.f0;
return new Tuple2<Integer, Integer>(i, i + 1);
}
}
public static class MySink implements SinkFunction<Tuple2<Integer, Integer>> {
private static final long serialVersionUID = 1L;
@Override
public void invoke(Tuple2<Integer, Integer> tuple) {
Integer k = tuple.getField(0);
Integer v = tuple.getField(1);
data.put(k, v);
}
}
@SuppressWarnings("unused")
private static final int SOURCE_PARALELISM = 1;
@Test
public void wrongJobGraph() {
StreamExecutionEnvironment env = StreamExecutionEnvironment.getExecutionEnvironment();
env.setParallelism(SOURCE_PARALELISM);
try {
env.fromCollection(null);
fail();
} catch (NullPointerException e) {
}
try {
env.fromElements();
fail();
} catch (IllegalArgumentException e) {
}
try {
env.generateSequence(-10, -30);
fail();
} catch (IllegalArgumentException e) {
}
try {
env.setBufferTimeout(-10);
fail();
} catch (IllegalArgumentException e) {
}
try {
env.generateSequence(1, 10).project(2);
fail();
} catch (RuntimeException e) {
}
}
private static class CoMap implements CoMapFunction<String, Long, String> {
private static final long serialVersionUID = 1L;
@Override
public String map1(String value) {
// System.out.println(value);
return value;
}
@Override
public String map2(Long value) {
// System.out.println(value);
return value.toString();
}
}
private static class SetSink implements SinkFunction<String> {
private static final long serialVersionUID = 1L;
public static Set<String> result = Collections.synchronizedSet(new HashSet<String>());
@Override
public void invoke(String value) {
result.add(value);
}
}
@Test
public void coTest() throws Exception {
StreamExecutionEnvironment env = StreamExecutionEnvironment.getExecutionEnvironment();
env.setParallelism(SOURCE_PARALELISM);
DataStream<String> fromStringElements = env.fromElements("aa", "bb", "cc");
DataStream<Long> generatedSequence = env.generateSequence(0, 3);
fromStringElements.connect(generatedSequence).map(new CoMap()).addSink(new SetSink());
env.execute();
HashSet<String> expectedSet = new HashSet<String>(Arrays.asList("aa", "bb", "cc", "0", "1",
"2", "3"));
assertEquals(expectedSet, SetSink.result);
}
@Test
public void runStream() throws Exception {
StreamExecutionEnvironment env = StreamExecutionEnvironment.getExecutionEnvironment();
env.setParallelism(SOURCE_PARALELISM);
env.addSource(new MySource()).setParallelism(SOURCE_PARALELISM).map(new MyTask())
.addSink(new MySink());
env.execute();
assertEquals(10, data.keySet().size());
for (Integer k : data.keySet()) {
assertEquals((Integer) (k + 1), data.get(k));
}
}
}
......@@ -27,17 +27,14 @@ import org.apache.flink.runtime.jobgraph.JobVertex;
import org.apache.flink.runtime.jobgraph.JobVertexID;
import org.apache.flink.streaming.api.datastream.DataStream;
import org.apache.flink.streaming.api.environment.StreamExecutionEnvironment;
import org.apache.flink.streaming.api.functions.sink.DiscardingSink;
import org.apache.flink.streaming.api.functions.sink.SinkFunction;
import org.apache.flink.streaming.api.functions.source.ParallelSourceFunction;
import org.apache.flink.streaming.api.graph.StreamGraph;
import org.apache.flink.streaming.api.graph.StreamNode;
import org.apache.flink.streaming.api.operators.ChainingStrategy;
import org.apache.flink.streaming.api.transformations.StreamTransformation;
import org.apache.flink.streaming.util.NoOpSink;
import org.junit.Test;
import java.lang.reflect.Field;
import java.util.Collection;
import java.util.HashMap;
import java.util.HashSet;
import java.util.List;
......@@ -54,6 +51,7 @@ import static org.junit.Assert.assertTrue;
* Tests the {@link StreamNode} hash assignment during translation from {@link StreamGraph} to
* {@link JobGraph} instances.
*/
@SuppressWarnings("serial")
public class StreamingJobGraphGeneratorNodeHashTest {
// ------------------------------------------------------------------------
......@@ -136,7 +134,7 @@ public class StreamingJobGraphGeneratorNodeHashTest {
env.disableOperatorChaining();
env.addSource(new NoOpSourceFunction(), "src").setParallelism(4)
.addSink(new NoOpSink<String>()).name("sink").setParallelism(4);
.addSink(new DiscardingSink<String>()).name("sink").setParallelism(4);
JobGraph jobGraph = env.getStreamGraph().getJobGraph();
......@@ -147,7 +145,7 @@ public class StreamingJobGraphGeneratorNodeHashTest {
env.disableOperatorChaining();
env.addSource(new NoOpSourceFunction(), "src").setParallelism(8)
.addSink(new NoOpSink<String>()).name("sink").setParallelism(4);
.addSink(new DiscardingSink<String>()).name("sink").setParallelism(4);
jobGraph = env.getStreamGraph().getJobGraph();
......@@ -158,7 +156,7 @@ public class StreamingJobGraphGeneratorNodeHashTest {
env.disableOperatorChaining();
env.addSource(new NoOpSourceFunction(), "src").setParallelism(4)
.addSink(new NoOpSink<String>()).name("sink").setParallelism(8);
.addSink(new DiscardingSink<String>()).name("sink").setParallelism(8);
jobGraph = env.getStreamGraph().getJobGraph();
......
......@@ -24,13 +24,12 @@ import org.apache.flink.streaming.api.environment.StreamExecutionEnvironment;
import org.apache.flink.streaming.api.functions.sink.SinkFunction;
import org.apache.flink.streaming.api.graph.StreamConfig;
import org.apache.flink.streaming.util.StreamingMultipleProgramsTestBase;
import org.junit.Test;
import static org.junit.Assert.*;
@SuppressWarnings("serial")
public class TranslationTest extends StreamingMultipleProgramsTestBase {
public class TranslationTest {
@Test
public void testCheckpointModeTranslation() {
......
......@@ -42,8 +42,8 @@ import org.apache.flink.streaming.api.windowing.triggers.EventTimeTrigger;
import org.apache.flink.streaming.api.windowing.triggers.Trigger;
import org.apache.flink.streaming.api.windowing.triggers.TriggerResult;
import org.apache.flink.streaming.api.windowing.windows.TimeWindow;
import org.apache.flink.streaming.util.StreamingMultipleProgramsTestBase;
import org.apache.flink.util.Collector;
import org.junit.Assert;
import org.junit.Test;
......@@ -56,7 +56,7 @@ import static org.junit.Assert.fail;
* {@link org.apache.flink.streaming.api.datastream.AllWindowedStream} instantiate
* the correct window operator.
*/
public class AllWindowTranslationTest extends StreamingMultipleProgramsTestBase {
public class AllWindowTranslationTest {
/**
* These tests ensure that the correct trigger is set when using event-time windows.
......@@ -265,8 +265,6 @@ public class AllWindowTranslationTest extends StreamingMultipleProgramsTestBase
}
fail("The fold call should fail.");
env.execute();
}
@Test
......@@ -314,9 +312,6 @@ public class AllWindowTranslationTest extends StreamingMultipleProgramsTestBase
}
fail("The trigger call should fail.");
env.execute();
}
// ------------------------------------------------------------------------
......
......@@ -35,8 +35,8 @@ import org.apache.flink.streaming.api.windowing.assigners.TumblingEventTimeWindo
import org.apache.flink.streaming.api.windowing.time.Time;
import org.apache.flink.streaming.api.windowing.triggers.EventTimeTrigger;
import org.apache.flink.streaming.api.windowing.windows.TimeWindow;
import org.apache.flink.streaming.util.StreamingMultipleProgramsTestBase;
import org.apache.flink.util.Collector;
import org.junit.Assert;
import org.junit.Ignore;
import org.junit.Test;
......@@ -48,7 +48,7 @@ import java.util.concurrent.TimeUnit;
* {@link WindowedStream} instantiate
* the correct window operator.
*/
public class TimeWindowTranslationTest extends StreamingMultipleProgramsTestBase {
public class TimeWindowTranslationTest {
/**
* These tests ensure that the fast aligned time windows operator is used if the
......
......@@ -44,8 +44,8 @@ import org.apache.flink.streaming.api.windowing.triggers.EventTimeTrigger;
import org.apache.flink.streaming.api.windowing.triggers.Trigger;
import org.apache.flink.streaming.api.windowing.triggers.TriggerResult;
import org.apache.flink.streaming.api.windowing.windows.TimeWindow;
import org.apache.flink.streaming.util.StreamingMultipleProgramsTestBase;
import org.apache.flink.util.Collector;
import org.junit.Assert;
import org.junit.Test;
......@@ -58,7 +58,8 @@ import static org.junit.Assert.fail;
* {@link WindowedStream} instantiate
* the correct window operator.
*/
public class WindowTranslationTest extends StreamingMultipleProgramsTestBase {
@SuppressWarnings("serial")
public class WindowTranslationTest {
/**
* .reduce() does not support RichReduceFunction, since the reduce function is used internally
......@@ -262,8 +263,6 @@ public class WindowTranslationTest extends StreamingMultipleProgramsTestBase {
}
fail("The fold call should fail.");
env.execute();
}
@Test
......@@ -317,8 +316,6 @@ public class WindowTranslationTest extends StreamingMultipleProgramsTestBase {
}
fail("The trigger call should fail.");
env.execute();
}
......
/*
* Licensed to the Apache Software Foundation (ASF) under one or more
* contributor license agreements. See the NOTICE file distributed with
* this work for additional information regarding copyright ownership.
* The ASF licenses this file to You under the Apache License, Version 2.0
* (the "License"); you may not use this file except in compliance with
* the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.flink.streaming.util;
import org.apache.flink.api.common.typeinfo.TypeInformation;
import org.apache.flink.api.java.typeutils.TypeExtractor;
import org.apache.flink.streaming.util.serialization.DeserializationSchema;
import org.apache.flink.streaming.util.serialization.SerializationSchema;
import org.apache.flink.streaming.util.serialization.SimpleStringSchema;
import org.apache.flink.test.testdata.WordCountData;
import org.apache.flink.util.NetUtils;
import org.junit.Assert;
import java.io.BufferedReader;
import java.io.InputStreamReader;
import java.net.ServerSocket;
import java.net.Socket;
import java.util.Arrays;
import java.util.HashSet;
import java.util.Set;
/**
* Test base for streaming programs relying on an open server socket to write to.
*/
public abstract class SocketOutputTestBase extends StreamingProgramTestBase {
protected static final String HOST = "localhost";
protected static Integer port;
protected Set<String> dataReadFromSocket = new HashSet<String>();
@Override
protected void preSubmit() throws Exception {
port = NetUtils.getAvailablePort();
temporarySocket = createLocalSocket(port);
}
@Override
protected void postSubmit() throws Exception {
Set<String> expectedData = new HashSet<String>(Arrays.asList(WordCountData.STREAMING_COUNTS_AS_TUPLES.split("\n")));
Assert.assertEquals(expectedData, dataReadFromSocket);
temporarySocket.close();
}
protected ServerSocket temporarySocket;
public ServerSocket createLocalSocket(int port) throws Exception {
ServerSocket serverSocket = new ServerSocket(port);
ServerThread st = new ServerThread(serverSocket);
st.start();
return serverSocket;
}
protected class ServerThread extends Thread {
private ServerSocket serverSocket;
private Thread t;
public ServerThread(ServerSocket serverSocket) {
this.serverSocket = serverSocket;
t = new Thread(this);
}
public void waitForAccept() throws Exception {
Socket socket = serverSocket.accept();
BufferedReader in = new BufferedReader(new InputStreamReader(socket.getInputStream()));
DeserializationSchema<String> schema = new SimpleStringSchema();
String rawData = in.readLine();
while (rawData != null){
String string = schema.deserialize(rawData.getBytes());
dataReadFromSocket.add(string);
rawData = in.readLine();
}
socket.close();
}
public void run() {
try {
waitForAccept();
} catch (Exception e) {
Assert.fail();
throw new RuntimeException(e);
}
}
@Override
public void start() {
t.start();
}
}
}
......@@ -98,15 +98,6 @@ under the License.
<type>test-jar</type>
</dependency>
<!-- To access streaming test utils -->
<dependency>
<groupId>org.apache.flink</groupId>
<artifactId>flink-streaming-java_2.10</artifactId>
<version>${project.version}</version>
<scope>test</scope>
<type>test-jar</type>
</dependency>
</dependencies>
<build>
......
/*
* Licensed to the Apache Software Foundation (ASF) under one or more
* contributor license agreements. See the NOTICE file distributed with
* this work for additional information regarding copyright ownership.
* The ASF licenses this file to You under the Apache License, Version 2.0
* (the "License"); you may not use this file except in compliance with
* the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.flink.streaming.scala.api;
import org.apache.flink.streaming.api.scala.OutputFormatTestPrograms;
import org.apache.flink.streaming.util.SocketOutputTestBase;
import org.apache.flink.test.testdata.WordCountData;
import org.junit.Ignore;
@Ignore
//This test sometimes fails most likely due to the behaviour
//of the socket. Disabled for now.
public class SocketOutputFormatITCase extends SocketOutputTestBase {
@Override
protected void testProgram() throws Exception {
OutputFormatTestPrograms.wordCountToSocket(WordCountData.TEXT, HOST, port);
}
}
......@@ -57,6 +57,13 @@ under the License.
<scope>compile</scope>
</dependency>
<dependency>
<groupId>org.apache.flink</groupId>
<artifactId>flink-streaming-java_2.10</artifactId>
<version>${project.version}</version>
<scope>compile</scope>
</dependency>
<dependency>
<groupId>junit</groupId>
<artifactId>junit</artifactId>
......
......@@ -22,6 +22,7 @@ import org.apache.flink.configuration.Configuration;
import org.apache.flink.test.util.AbstractTestBase;
import org.apache.flink.test.util.ForkableFlinkMiniCluster;
import org.apache.flink.test.util.TestBaseUtils;
import org.junit.AfterClass;
import org.junit.BeforeClass;
......
......@@ -542,7 +542,7 @@ under the License.
</goals>
</pluginExecutionFilter>
<action>
<ignore></ignore>
<ignore/>
</action>
</pluginExecution>
</pluginExecutions>
......
......@@ -16,7 +16,7 @@
* limitations under the License.
*/
package org.apache.flink.streaming.api;
package org.apache.flink.test.streaming.api;
import org.apache.flink.api.common.functions.FoldFunction;
import org.apache.flink.api.common.functions.MapFunction;
......@@ -30,6 +30,7 @@ import org.apache.flink.streaming.api.datastream.SplitStream;
import org.apache.flink.streaming.api.environment.StreamExecutionEnvironment;
import org.apache.flink.streaming.api.functions.source.SourceFunction;
import org.apache.flink.streaming.util.StreamingMultipleProgramsTestBase;
import org.junit.After;
import org.junit.Before;
import org.junit.Rule;
......
......@@ -15,7 +15,7 @@
* limitations under the License.
*/
package org.apache.flink.streaming.api.outputformat;
package org.apache.flink.test.streaming.api.outputformat;
import org.apache.flink.api.common.functions.FlatMapFunction;
import org.apache.flink.api.java.tuple.Tuple2;
......
......@@ -15,7 +15,7 @@
* limitations under the License.
*/
package org.apache.flink.streaming.api.outputformat;
package org.apache.flink.test.streaming.api.outputformat;
import org.apache.flink.api.java.tuple.Tuple2;
import org.apache.flink.streaming.api.datastream.DataStream;
......
......@@ -15,22 +15,22 @@
* limitations under the License.
*/
package org.apache.flink.streaming.api;
import static org.junit.Assert.*;
package org.apache.flink.test.streaming.runtime;
import org.apache.flink.api.common.functions.RichMapFunction;
import org.apache.flink.api.common.functions.RuntimeContext;
import org.apache.flink.configuration.Configuration;
import org.apache.flink.streaming.api.environment.StreamExecutionEnvironment;
import org.apache.flink.streaming.api.functions.sink.DiscardingSink;
import org.apache.flink.streaming.api.functions.source.RichParallelSourceFunction;
import org.apache.flink.streaming.util.NoOpSink;
import org.apache.flink.streaming.util.StreamingMultipleProgramsTestBase;
import org.junit.Test;
import static org.junit.Assert.assertNotEquals;
@SuppressWarnings("serial")
public class ChainedRuntimeContextTest extends StreamingMultipleProgramsTestBase {
public class ChainedRuntimeContextITCase extends StreamingMultipleProgramsTestBase {
private static RuntimeContext srcContext;
private static RuntimeContext mapContext;
......@@ -39,7 +39,7 @@ public class ChainedRuntimeContextTest extends StreamingMultipleProgramsTestBase
StreamExecutionEnvironment env = StreamExecutionEnvironment.getExecutionEnvironment();
env.setParallelism(1);
env.addSource(new TestSource()).map(new TestMap()).addSink(new NoOpSink<Integer>());
env.addSource(new TestSource()).map(new TestMap()).addSink(new DiscardingSink<Integer>());
env.execute();
assertNotEquals(srcContext, mapContext);
......
......@@ -15,7 +15,7 @@
* limitations under the License.
*/
package org.apache.flink.streaming.runtime.operators.windowing;
package org.apache.flink.test.streaming.runtime;
import org.apache.flink.api.common.functions.CoGroupFunction;
import org.apache.flink.api.common.functions.JoinFunction;
......@@ -33,7 +33,6 @@ import org.apache.flink.streaming.api.windowing.assigners.TumblingEventTimeWindo
import org.apache.flink.streaming.api.windowing.time.Time;
import org.apache.flink.streaming.util.StreamingMultipleProgramsTestBase;
import org.apache.flink.util.Collector;
import org.junit.Assert;
import org.junit.Test;
......
......@@ -15,13 +15,7 @@
* limitations under the License.
*/
package org.apache.flink.streaming.api;
import static org.junit.Assert.assertEquals;
import java.util.Arrays;
import java.util.Collections;
import java.util.List;
package org.apache.flink.test.streaming.runtime;
import org.apache.flink.api.common.functions.FilterFunction;
import org.apache.flink.api.common.functions.MapFunction;
......@@ -31,11 +25,17 @@ import org.apache.flink.streaming.api.datastream.DataStream;
import org.apache.flink.streaming.api.environment.StreamExecutionEnvironment;
import org.apache.flink.streaming.api.functions.co.CoFlatMapFunction;
import org.apache.flink.streaming.util.StreamingMultipleProgramsTestBase;
import org.apache.flink.streaming.util.TestListResultSink;
import org.apache.flink.test.streaming.runtime.util.TestListResultSink;
import org.apache.flink.util.Collector;
import org.junit.Test;
import java.util.Arrays;
import java.util.Collections;
import java.util.List;
import static org.junit.Assert.assertEquals;
@SuppressWarnings("serial")
public class CoStreamITCase extends StreamingMultipleProgramsTestBase {
......
......@@ -15,12 +15,11 @@
* limitations under the License.
*/
package org.apache.flink.streaming.api;
package org.apache.flink.test.streaming.runtime;
import org.apache.flink.api.common.functions.FlatMapFunction;
import org.apache.flink.streaming.api.datastream.DataStream;
import org.apache.flink.streaming.api.environment.StreamExecutionEnvironment;
import org.apache.flink.streaming.api.functions.source.SourceFunction;
import org.apache.flink.streaming.util.StreamingMultipleProgramsTestBase;
import org.apache.flink.util.Collector;
import org.junit.Test;
......
......@@ -15,26 +15,23 @@
* limitations under the License.
*/
package org.apache.flink.streaming.api.collector;
import static org.junit.Assert.assertEquals;
import java.io.IOException;
import java.util.ArrayList;
import java.util.Arrays;
import java.util.HashMap;
import java.util.List;
import java.util.Map;
package org.apache.flink.test.streaming.runtime;
import org.apache.flink.streaming.api.collector.selector.OutputSelector;
import org.apache.flink.streaming.api.datastream.SplitStream;
import org.apache.flink.streaming.api.environment.StreamExecutionEnvironment;
import org.apache.flink.streaming.api.functions.sink.SinkFunction;
import org.apache.flink.streaming.util.StreamingMultipleProgramsTestBase;
import org.apache.flink.streaming.util.TestListResultSink;
import org.apache.flink.test.streaming.runtime.util.TestListResultSink;
import org.junit.Test;
public class DirectedOutputTest extends StreamingMultipleProgramsTestBase {
import java.util.ArrayList;
import java.util.Arrays;
import java.util.List;
import static org.junit.Assert.assertEquals;
public class DirectedOutputITCase extends StreamingMultipleProgramsTestBase {
private static final String TEN = "ten";
private static final String ODD = "odd";
......@@ -66,32 +63,6 @@ public class DirectedOutputTest extends StreamingMultipleProgramsTestBase {
}
}
static final class ListSink implements SinkFunction<Long> {
private static final long serialVersionUID = 1L;
private String name;
private transient List<Long> list;
public ListSink(String name) {
this.name = name;
}
@Override
public void invoke(Long value) {
list.add(value);
}
private void readObject(java.io.ObjectInputStream in) throws IOException,
ClassNotFoundException {
in.defaultReadObject();
outputs.put(name, new ArrayList<Long>());
this.list = outputs.get(name);
}
}
private static Map<String, List<Long>> outputs = new HashMap<String, List<Long>>();
@Test
public void outputSelectorTest() throws Exception {
StreamExecutionEnvironment env = StreamExecutionEnvironment.getExecutionEnvironment();
......
......@@ -15,7 +15,7 @@
* limitations under the License.
*/
package org.apache.flink.streaming.api;
package org.apache.flink.test.streaming.runtime;
import org.apache.flink.api.common.InvalidProgramException;
import org.apache.flink.api.common.functions.MapFunction;
......@@ -24,6 +24,7 @@ import org.apache.flink.api.java.functions.KeySelector;
import org.apache.flink.api.java.tuple.Tuple2;
import org.apache.flink.runtime.jobgraph.JobGraph;
import org.apache.flink.runtime.jobgraph.JobVertex;
import org.apache.flink.streaming.api.CheckpointingMode;
import org.apache.flink.streaming.api.datastream.DataStream;
import org.apache.flink.streaming.api.datastream.DataStreamSink;
import org.apache.flink.streaming.api.datastream.IterativeStream;
......@@ -41,13 +42,15 @@ import org.apache.flink.streaming.runtime.partitioner.BroadcastPartitioner;
import org.apache.flink.streaming.runtime.partitioner.ForwardPartitioner;
import org.apache.flink.streaming.runtime.partitioner.RebalancePartitioner;
import org.apache.flink.streaming.runtime.partitioner.ShufflePartitioner;
import org.apache.flink.streaming.util.EvenOddOutputSelector;
import org.apache.flink.streaming.util.NoOpIntMap;
import org.apache.flink.streaming.util.ReceiveCheckNoOpSink;
import org.apache.flink.streaming.util.StreamingMultipleProgramsTestBase;
import org.apache.flink.test.streaming.runtime.util.EvenOddOutputSelector;
import org.apache.flink.test.streaming.runtime.util.NoOpIntMap;
import org.apache.flink.test.streaming.runtime.util.ReceiveCheckNoOpSink;
import org.apache.flink.util.Collector;
import org.apache.flink.util.MathUtils;
import org.junit.Test;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
......@@ -61,9 +64,9 @@ import static org.junit.Assert.assertTrue;
import static org.junit.Assert.fail;
@SuppressWarnings({ "unchecked", "unused", "serial" })
public class IterateTest extends StreamingMultipleProgramsTestBase {
public class IterateITCase extends StreamingMultipleProgramsTestBase {
private static final Logger LOG = LoggerFactory.getLogger(IterateTest.class);
private static final Logger LOG = LoggerFactory.getLogger(IterateITCase.class);
private static boolean iterated[];
......
......@@ -15,23 +15,23 @@
* limitations under the License.
*/
package org.apache.flink.streaming.api;
import static org.junit.Assert.assertEquals;
import java.util.ArrayList;
import java.util.Arrays;
import java.util.List;
package org.apache.flink.test.streaming.runtime;
import org.apache.flink.streaming.api.collector.selector.OutputSelector;
import org.apache.flink.streaming.api.datastream.DataStream;
import org.apache.flink.streaming.api.environment.StreamExecutionEnvironment;
import org.apache.flink.streaming.util.StreamingMultipleProgramsTestBase;
import org.apache.flink.streaming.util.TestListResultSink;
import org.apache.flink.test.streaming.runtime.util.TestListResultSink;
import org.junit.Test;
public class OutputSplitterTest extends StreamingMultipleProgramsTestBase {
import java.util.ArrayList;
import java.util.Arrays;
import java.util.List;
import static org.junit.Assert.assertEquals;
public class OutputSplitterITCase extends StreamingMultipleProgramsTestBase {
private static ArrayList<Integer> expectedSplitterResult = new ArrayList<Integer>();
......
......@@ -15,15 +15,7 @@
* limitations under the License.
*/
package org.apache.flink.streaming.api;
import static org.junit.Assert.assertEquals;
import static org.junit.Assert.fail;
import java.util.Arrays;
import java.util.HashMap;
import java.util.HashSet;
import java.util.List;
package org.apache.flink.test.streaming.runtime;
import org.apache.flink.api.common.functions.MapFunction;
import org.apache.flink.api.common.functions.Partitioner;
......@@ -34,16 +26,25 @@ import org.apache.flink.api.java.tuple.Tuple2;
import org.apache.flink.configuration.Configuration;
import org.apache.flink.streaming.api.datastream.DataStream;
import org.apache.flink.streaming.api.environment.StreamExecutionEnvironment;
import org.apache.flink.streaming.util.NoOpIntMap;
import org.apache.flink.streaming.util.StreamingMultipleProgramsTestBase;
import org.apache.flink.streaming.util.TestListResultSink;
import org.apache.flink.test.streaming.runtime.util.NoOpIntMap;
import org.apache.flink.test.streaming.runtime.util.TestListResultSink;
import org.junit.Test;
import java.util.Arrays;
import java.util.HashMap;
import java.util.HashSet;
import java.util.List;
import static org.junit.Assert.assertEquals;
import static org.junit.Assert.fail;
/**
* IT case that tests the different stream partitioning schemes.
*/
public class PartitionerTest extends StreamingMultipleProgramsTestBase {
@SuppressWarnings("serial")
public class PartitionerITCase extends StreamingMultipleProgramsTestBase {
@Test(expected = UnsupportedOperationException.class)
public void testForwardFailsLowToHighParallelism() throws Exception {
......
......@@ -15,14 +15,7 @@
* limitations under the License.
*/
package org.apache.flink.streaming.api.operators.co;
import static org.junit.Assert.assertEquals;
import java.util.ArrayList;
import java.util.Arrays;
import java.util.Collections;
import java.util.List;
package org.apache.flink.test.streaming.runtime;
import org.apache.flink.api.common.functions.FlatMapFunction;
import org.apache.flink.api.common.functions.MapFunction;
......@@ -31,19 +24,24 @@ import org.apache.flink.streaming.api.datastream.DataStream;
import org.apache.flink.streaming.api.environment.StreamExecutionEnvironment;
import org.apache.flink.streaming.api.functions.co.CoMapFunction;
import org.apache.flink.streaming.util.StreamingMultipleProgramsTestBase;
import org.apache.flink.streaming.util.TestListResultSink;
import org.apache.flink.test.streaming.runtime.util.TestListResultSink;
import org.apache.flink.util.Collector;
import org.junit.Test;
public class SelfConnectionTest extends StreamingMultipleProgramsTestBase {
import java.util.Arrays;
import java.util.Collections;
import java.util.List;
import static org.junit.Assert.assertEquals;
private static List<String> expected;
public class SelfConnectionITCase extends StreamingMultipleProgramsTestBase {
/**
* We connect two different data streams in a chain to a CoMap.
*/
@Test
public void differentDataStreamSameChain() {
public void differentDataStreamSameChain() throws Exception {
TestListResultSink<String> resultSink = new TestListResultSink<String>();
......@@ -76,15 +74,9 @@ public class SelfConnectionTest extends StreamingMultipleProgramsTestBase {
}
}).addSink(resultSink);
try {
env.execute();
} catch (Exception e) {
e.printStackTrace();
}
expected = new ArrayList<String>();
env.execute();
expected.addAll(Arrays.asList("x 1", "x 3", "x 5", "2", "4", "6"));
List<String> expected = Arrays.asList("x 1", "x 3", "x 5", "2", "4", "6");
List<String> result = resultSink.getResult();
......@@ -132,7 +124,7 @@ public class SelfConnectionTest extends StreamingMultipleProgramsTestBase {
@Override
public Long map(Integer value) throws Exception {
return Long.valueOf(value + 1);
return (long) (value + 1);
}
}).keyBy(new KeySelector<Long, Integer>() {
......@@ -166,10 +158,7 @@ public class SelfConnectionTest extends StreamingMultipleProgramsTestBase {
e.printStackTrace();
}
expected = new ArrayList<String>();
expected.addAll(Arrays.asList("x 1", "x 3", "x 5", "2", "4", "6"));
List<String> expected = Arrays.asList("x 1", "x 3", "x 5", "2", "4", "6");
List<String> result = resultSink.getResult();
Collections.sort(expected);
......
......@@ -16,11 +16,11 @@
* limitations under the License.
*/
package org.apache.flink.streaming.runtime.state;
package org.apache.flink.test.streaming.runtime;
import org.apache.flink.api.common.functions.RichMapFunction;
import org.apache.flink.api.common.state.FoldingState;
import org.apache.flink.api.common.restartstrategy.RestartStrategies;
import org.apache.flink.api.common.state.FoldingState;
import org.apache.flink.api.common.state.FoldingStateDescriptor;
import org.apache.flink.api.common.state.ListState;
import org.apache.flink.api.common.state.ListStateDescriptor;
......@@ -37,7 +37,6 @@ import org.apache.flink.runtime.state.AbstractStateBackend;
import org.apache.flink.runtime.state.StateHandle;
import org.apache.flink.streaming.api.environment.StreamExecutionEnvironment;
import org.apache.flink.streaming.util.StreamingMultipleProgramsTestBase;
import org.junit.Test;
import java.io.Serializable;
......
......@@ -16,7 +16,7 @@
* limitations under the License.
*/
package org.apache.flink.streaming.runtime.tasks;
package org.apache.flink.test.streaming.runtime;
import org.apache.flink.api.common.typeinfo.BasicTypeInfo;
import org.apache.flink.runtime.client.JobExecutionException;
......@@ -31,6 +31,7 @@ import org.apache.flink.streaming.api.operators.TwoInputStreamOperator;
import org.apache.flink.streaming.api.watermark.Watermark;
import org.apache.flink.streaming.runtime.operators.Triggerable;
import org.apache.flink.streaming.runtime.streamrecord.StreamRecord;
import org.apache.flink.streaming.runtime.tasks.TimerException;
import org.apache.flink.streaming.util.StreamingMultipleProgramsTestBase;
import org.junit.Assert;
......
......@@ -16,7 +16,7 @@
* limitations under the License.
*/
package org.apache.flink.streaming.timestamp;
package org.apache.flink.test.streaming.runtime;
import org.apache.flink.api.common.JobID;
import org.apache.flink.api.common.functions.MapFunction;
......@@ -30,6 +30,7 @@ import org.apache.flink.core.testutils.MultiShotLatch;
import org.apache.flink.streaming.api.TimeCharacteristic;
import org.apache.flink.streaming.api.datastream.DataStream;
import org.apache.flink.streaming.api.environment.StreamExecutionEnvironment;
import org.apache.flink.streaming.api.functions.sink.DiscardingSink;
import org.apache.flink.streaming.api.functions.timestamps.AscendingTimestampExtractor;
import org.apache.flink.streaming.api.functions.AssignerWithPeriodicWatermarks;
import org.apache.flink.streaming.api.functions.AssignerWithPunctuatedWatermarks;
......@@ -42,10 +43,9 @@ import org.apache.flink.streaming.api.watermark.Watermark;
import org.apache.flink.streaming.api.windowing.assigners.TumblingEventTimeWindows;
import org.apache.flink.streaming.api.windowing.time.Time;
import org.apache.flink.streaming.runtime.streamrecord.StreamRecord;
import org.apache.flink.streaming.util.NoOpSink;
import org.apache.flink.test.util.ForkableFlinkMiniCluster;
import org.apache.flink.util.TestLogger;
import org.junit.AfterClass;
import org.junit.Assert;
import org.junit.Before;
......@@ -145,7 +145,7 @@ public class TimestampITCase extends TestLogger {
.map(new IdentityMap())
.connect(source2).map(new IdentityCoMap())
.transform("Custom Operator", BasicTypeInfo.INT_TYPE_INFO, new CustomOperator(true))
.addSink(new NoOpSink<Integer>());
.addSink(new DiscardingSink<Integer>());
env.execute();
......@@ -195,7 +195,7 @@ public class TimestampITCase extends TestLogger {
.map(new IdentityMap())
.connect(source2).map(new IdentityCoMap())
.transform("Custom Operator", BasicTypeInfo.INT_TYPE_INFO, new CustomOperator(true))
.addSink(new NoOpSink<Integer>());
.addSink(new DiscardingSink<Integer>());
new Thread("stopper") {
@Override
......@@ -270,7 +270,7 @@ public class TimestampITCase extends TestLogger {
.map(new IdentityMap())
.connect(source2).map(new IdentityCoMap())
.transform("Custom Operator", BasicTypeInfo.INT_TYPE_INFO, new TimestampCheckingOperator())
.addSink(new NoOpSink<Integer>());
.addSink(new DiscardingSink<Integer>());
env.execute();
......@@ -297,7 +297,7 @@ public class TimestampITCase extends TestLogger {
.map(new IdentityMap())
.connect(source2).map(new IdentityCoMap())
.transform("Custom Operator", BasicTypeInfo.INT_TYPE_INFO, new DisabledTimestampCheckingOperator())
.addSink(new NoOpSink<Integer>());
.addSink(new DiscardingSink<Integer>());
env.execute();
}
......
......@@ -15,7 +15,7 @@
* limitations under the License.
*/
package org.apache.flink.streaming.runtime.operators.windowing;
package org.apache.flink.test.streaming.runtime;
import org.apache.flink.api.common.functions.FoldFunction;
import org.apache.flink.api.java.tuple.Tuple2;
......@@ -29,7 +29,6 @@ import org.apache.flink.streaming.api.watermark.Watermark;
import org.apache.flink.streaming.api.windowing.assigners.TumblingEventTimeWindows;
import org.apache.flink.streaming.api.windowing.time.Time;
import org.apache.flink.streaming.util.StreamingMultipleProgramsTestBase;
import org.junit.Assert;
import org.junit.Test;
......
/*
/**
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
......@@ -7,7 +7,7 @@
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
......@@ -15,19 +15,17 @@
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.flink.test.streaming.runtime.util;
package org.apache.flink.streaming.connectors.kafka.testutils;
import org.apache.flink.streaming.api.functions.sink.SinkFunction;
import org.apache.flink.streaming.api.collector.selector.OutputSelector;
/**
* Sink function that discards data.
* @param <T> The type of the function.
*/
public class DiscardingSink<T> implements SinkFunction<T> {
import java.util.Collections;
private static final long serialVersionUID = 2777597566520109843L;
public class EvenOddOutputSelector implements OutputSelector<Integer> {
private static final long serialVersionUID = 1L;
@Override
public void invoke(T value) {}
public Iterable<String> select(Integer value) {
return value % 2 == 0 ? Collections.singleton("even") : Collections.singleton("odd");
}
}
......@@ -15,12 +15,14 @@
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.flink.streaming.util;
package org.apache.flink.test.streaming.runtime.util;
import org.apache.flink.streaming.api.functions.sink.RichSinkFunction;
import org.apache.flink.api.common.functions.MapFunction;
public final class NoOpSink<T> extends RichSinkFunction<T> {
public void invoke(T tuple) {
public class NoOpIntMap implements MapFunction<Integer, Integer> {
private static final long serialVersionUID = 1L;
public Integer map(Integer value) throws Exception {
return value;
}
}
......@@ -15,7 +15,7 @@
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.flink.streaming.util;
package org.apache.flink.test.streaming.runtime.util;
import org.apache.flink.configuration.Configuration;
import org.apache.flink.streaming.api.functions.sink.RichSinkFunction;
......
......@@ -15,15 +15,15 @@
* limitations under the License.
*/
package org.apache.flink.streaming.util;
package org.apache.flink.test.streaming.runtime.util;
import org.apache.flink.configuration.Configuration;
import org.apache.flink.streaming.api.functions.sink.RichSinkFunction;
import java.util.ArrayList;
import java.util.List;
import java.util.TreeSet;
import org.apache.flink.configuration.Configuration;
import org.apache.flink.streaming.api.functions.sink.RichSinkFunction;
public class TestListResultSink<T> extends RichSinkFunction<T> {
private static final long serialVersionUID = 1L;
......
......@@ -15,7 +15,7 @@
* limitations under the License.
*/
package org.apache.flink.streaming.util;
package org.apache.flink.test.streaming.runtime.util;
import java.util.ArrayList;
import java.util.Collections;
......
Markdown is supported
0% .
You are about to add 0 people to the discussion. Proceed with caution.
先完成此消息的编辑!
想要评论请 注册