[
https://issues.apache.org/jira/browse/FLINK-10800?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=16677924#comment-16677924
]
ASF GitHub Bot commented on FLINK-10800:
----------------------------------------
pnowojski closed pull request #7031: [FLINK-10800][network][test] Abstract
StreamPartitionerTest for common codes
URL: https://github.com/apache/flink/pull/7031
This is a PR merged from a forked repository.
As GitHub hides the original diff on merge, it is displayed below for
the sake of provenance:
As this is a foreign pull request (from a fork), the diff is supplied
below (as it won't show otherwise due to GitHub magic):
diff --git
a/flink-streaming-java/src/test/java/org/apache/flink/streaming/runtime/partitioner/BroadcastPartitionerTest.java
b/flink-streaming-java/src/test/java/org/apache/flink/streaming/runtime/partitioner/BroadcastPartitionerTest.java
index 307700530dd..63a45fde064 100644
---
a/flink-streaming-java/src/test/java/org/apache/flink/streaming/runtime/partitioner/BroadcastPartitionerTest.java
+++
b/flink-streaming-java/src/test/java/org/apache/flink/streaming/runtime/partitioner/BroadcastPartitionerTest.java
@@ -35,15 +35,14 @@
private BroadcastPartitioner<Tuple> broadcastPartitioner2;
private BroadcastPartitioner<Tuple> broadcastPartitioner3;
- private StreamRecord<Tuple> streamRecord = new
StreamRecord<Tuple>(null);
- private SerializationDelegate<StreamRecord<Tuple>> sd = new
SerializationDelegate<StreamRecord<Tuple>>(null);
+ private StreamRecord<Tuple> streamRecord = new StreamRecord<>(null);
+ private SerializationDelegate<StreamRecord<Tuple>>
serializationDelegate = new SerializationDelegate<>(null);
@Before
public void setPartitioner() {
- broadcastPartitioner1 = new BroadcastPartitioner<Tuple>();
- broadcastPartitioner2 = new BroadcastPartitioner<Tuple>();
- broadcastPartitioner3 = new BroadcastPartitioner<Tuple>();
-
+ broadcastPartitioner1 = new BroadcastPartitioner<>();
+ broadcastPartitioner2 = new BroadcastPartitioner<>();
+ broadcastPartitioner3 = new BroadcastPartitioner<>();
}
@Test
@@ -51,9 +50,11 @@ public void testSelectChannels() {
int[] first = new int[] { 0 };
int[] second = new int[] { 0, 1 };
int[] sixth = new int[] { 0, 1, 2, 3, 4, 5 };
- sd.setInstance(streamRecord);
- assertArrayEquals(first,
broadcastPartitioner1.selectChannels(sd, 1));
- assertArrayEquals(second,
broadcastPartitioner2.selectChannels(sd, 2));
- assertArrayEquals(sixth,
broadcastPartitioner3.selectChannels(sd, 6));
+
+ serializationDelegate.setInstance(streamRecord);
+
+ assertArrayEquals(first,
broadcastPartitioner1.selectChannels(serializationDelegate, 1));
+ assertArrayEquals(second,
broadcastPartitioner2.selectChannels(serializationDelegate, 2));
+ assertArrayEquals(sixth,
broadcastPartitioner3.selectChannels(serializationDelegate, 6));
}
}
diff --git
a/flink-streaming-java/src/test/java/org/apache/flink/streaming/runtime/partitioner/ForwardPartitionerTest.java
b/flink-streaming-java/src/test/java/org/apache/flink/streaming/runtime/partitioner/ForwardPartitionerTest.java
index 2ecf17b9e1d..9b84b1238b1 100644
---
a/flink-streaming-java/src/test/java/org/apache/flink/streaming/runtime/partitioner/ForwardPartitionerTest.java
+++
b/flink-streaming-java/src/test/java/org/apache/flink/streaming/runtime/partitioner/ForwardPartitionerTest.java
@@ -18,42 +18,23 @@
package org.apache.flink.streaming.runtime.partitioner;
import org.apache.flink.api.java.tuple.Tuple;
-import org.apache.flink.runtime.plugable.SerializationDelegate;
-import org.apache.flink.streaming.runtime.streamrecord.StreamRecord;
-import org.junit.Before;
import org.junit.Test;
-import static org.junit.Assert.assertEquals;
-
/**
* Tests for {@link ForwardPartitioner}.
*/
-public class ForwardPartitionerTest {
-
- private ForwardPartitioner<Tuple> forwardPartitioner;
- private StreamRecord<Tuple> streamRecord = new
StreamRecord<Tuple>(null);
- private SerializationDelegate<StreamRecord<Tuple>> sd = new
SerializationDelegate<StreamRecord<Tuple>>(
- null);
+public class ForwardPartitionerTest extends StreamPartitionerTest {
- @Before
- public void setPartitioner() {
- forwardPartitioner = new ForwardPartitioner<Tuple>();
- }
-
- @Test
- public void testSelectChannelsLength() {
- sd.setInstance(streamRecord);
- assertEquals(1, forwardPartitioner.selectChannels(sd,
1).length);
- assertEquals(1, forwardPartitioner.selectChannels(sd,
2).length);
- assertEquals(1, forwardPartitioner.selectChannels(sd,
1024).length);
+ @Override
+ public StreamPartitioner<Tuple> createPartitioner() {
+ return new ForwardPartitioner<>();
}
@Test
public void testSelectChannelsInterval() {
- sd.setInstance(streamRecord);
- assertEquals(0, forwardPartitioner.selectChannels(sd, 1)[0]);
- assertEquals(0, forwardPartitioner.selectChannels(sd, 2)[0]);
- assertEquals(0, forwardPartitioner.selectChannels(sd, 1024)[0]);
+ assertSelectedChannel(0, 1);
+ assertSelectedChannel(0, 2);
+ assertSelectedChannel(0, 1024);
}
}
diff --git
a/flink-streaming-java/src/test/java/org/apache/flink/streaming/runtime/partitioner/GlobalPartitionerTest.java
b/flink-streaming-java/src/test/java/org/apache/flink/streaming/runtime/partitioner/GlobalPartitionerTest.java
index 5d023c84bb4..b382f3e6ea8 100644
---
a/flink-streaming-java/src/test/java/org/apache/flink/streaming/runtime/partitioner/GlobalPartitionerTest.java
+++
b/flink-streaming-java/src/test/java/org/apache/flink/streaming/runtime/partitioner/GlobalPartitionerTest.java
@@ -18,37 +18,23 @@
package org.apache.flink.streaming.runtime.partitioner;
import org.apache.flink.api.java.tuple.Tuple;
-import org.apache.flink.runtime.plugable.SerializationDelegate;
-import org.apache.flink.streaming.runtime.streamrecord.StreamRecord;
-import org.junit.Before;
import org.junit.Test;
-import static org.junit.Assert.assertArrayEquals;
-
/**
* Tests for {@link GlobalPartitioner}.
*/
-public class GlobalPartitionerTest {
-
- private GlobalPartitioner<Tuple> globalPartitioner;
- private StreamRecord<Tuple> streamRecord = new
StreamRecord<Tuple>(null);
- private SerializationDelegate<StreamRecord<Tuple>> sd = new
SerializationDelegate<StreamRecord<Tuple>>(
- null);
+public class GlobalPartitionerTest extends StreamPartitionerTest {
- @Before
- public void setPartitioner() {
- globalPartitioner = new GlobalPartitioner<Tuple>();
+ @Override
+ public StreamPartitioner<Tuple> createPartitioner() {
+ return new GlobalPartitioner<>();
}
@Test
public void testSelectChannels() {
- int[] result = new int[] { 0 };
-
- sd.setInstance(streamRecord);
-
- assertArrayEquals(result, globalPartitioner.selectChannels(sd,
1));
- assertArrayEquals(result, globalPartitioner.selectChannels(sd,
2));
- assertArrayEquals(result, globalPartitioner.selectChannels(sd,
1024));
+ assertSelectedChannel(0, 1);
+ assertSelectedChannel(0, 2);
+ assertSelectedChannel(0, 1024);
}
}
diff --git
a/flink-streaming-java/src/test/java/org/apache/flink/streaming/runtime/partitioner/KeyGroupStreamPartitionerTest.java
b/flink-streaming-java/src/test/java/org/apache/flink/streaming/runtime/partitioner/KeyGroupStreamPartitionerTest.java
index a57e6f47d70..07b57217221 100644
---
a/flink-streaming-java/src/test/java/org/apache/flink/streaming/runtime/partitioner/KeyGroupStreamPartitionerTest.java
+++
b/flink-streaming-java/src/test/java/org/apache/flink/streaming/runtime/partitioner/KeyGroupStreamPartitionerTest.java
@@ -35,14 +35,14 @@
public class KeyGroupStreamPartitionerTest extends TestLogger {
private KeyGroupStreamPartitioner<Tuple2<String, Integer>, String>
keyGroupPartitioner;
- private StreamRecord<Tuple2<String, Integer>> streamRecord1 = new
StreamRecord<Tuple2<String, Integer>>(new Tuple2<String, Integer>("test", 0));
- private StreamRecord<Tuple2<String, Integer>> streamRecord2 = new
StreamRecord<Tuple2<String, Integer>>(new Tuple2<String, Integer>("test", 42));
- private SerializationDelegate<StreamRecord<Tuple2<String, Integer>>>
sd1 = new SerializationDelegate<StreamRecord<Tuple2<String, Integer>>>(null);
- private SerializationDelegate<StreamRecord<Tuple2<String, Integer>>>
sd2 = new SerializationDelegate<StreamRecord<Tuple2<String, Integer>>>(null);
+ private StreamRecord<Tuple2<String, Integer>> streamRecord1 = new
StreamRecord<>(new Tuple2<>("test", 0));
+ private StreamRecord<Tuple2<String, Integer>> streamRecord2 = new
StreamRecord<>(new Tuple2<>("test", 42));
+ private SerializationDelegate<StreamRecord<Tuple2<String, Integer>>>
serializationDelegate1 = new SerializationDelegate<>(null);
+ private SerializationDelegate<StreamRecord<Tuple2<String, Integer>>>
serializationDelegate2 = new SerializationDelegate<>(null);
@Before
public void setPartitioner() {
- keyGroupPartitioner = new
KeyGroupStreamPartitioner<Tuple2<String, Integer>, String>(new
KeySelector<Tuple2<String, Integer>, String>() {
+ keyGroupPartitioner = new KeyGroupStreamPartitioner<>(new
KeySelector<Tuple2<String, Integer>, String>() {
private static final long serialVersionUID = 1L;
@@ -56,22 +56,23 @@ public String getKey(Tuple2<String, Integer> value) throws
Exception {
@Test
public void testSelectChannelsLength() {
- sd1.setInstance(streamRecord1);
- assertEquals(1, keyGroupPartitioner.selectChannels(sd1,
1).length);
- assertEquals(1, keyGroupPartitioner.selectChannels(sd1,
2).length);
- assertEquals(1, keyGroupPartitioner.selectChannels(sd1,
1024).length);
+ serializationDelegate1.setInstance(streamRecord1);
+
+ assertEquals(1,
keyGroupPartitioner.selectChannels(serializationDelegate1, 1).length);
+ assertEquals(1,
keyGroupPartitioner.selectChannels(serializationDelegate1, 2).length);
+ assertEquals(1,
keyGroupPartitioner.selectChannels(serializationDelegate1, 1024).length);
}
@Test
public void testSelectChannelsGrouping() {
- sd1.setInstance(streamRecord1);
- sd2.setInstance(streamRecord2);
+ serializationDelegate1.setInstance(streamRecord1);
+ serializationDelegate2.setInstance(streamRecord2);
- assertArrayEquals(keyGroupPartitioner.selectChannels(sd1, 1),
- keyGroupPartitioner.selectChannels(sd2, 1));
- assertArrayEquals(keyGroupPartitioner.selectChannels(sd1, 2),
- keyGroupPartitioner.selectChannels(sd2, 2));
- assertArrayEquals(keyGroupPartitioner.selectChannels(sd1, 1024),
- keyGroupPartitioner.selectChannels(sd2, 1024));
+
assertArrayEquals(keyGroupPartitioner.selectChannels(serializationDelegate1, 1),
+
keyGroupPartitioner.selectChannels(serializationDelegate2, 1));
+
assertArrayEquals(keyGroupPartitioner.selectChannels(serializationDelegate1, 2),
+
keyGroupPartitioner.selectChannels(serializationDelegate2, 2));
+
assertArrayEquals(keyGroupPartitioner.selectChannels(serializationDelegate1,
1024),
+
keyGroupPartitioner.selectChannels(serializationDelegate2, 1024));
}
}
diff --git
a/flink-streaming-java/src/test/java/org/apache/flink/streaming/runtime/partitioner/RebalancePartitionerTest.java
b/flink-streaming-java/src/test/java/org/apache/flink/streaming/runtime/partitioner/RebalancePartitionerTest.java
index 0cf022b3f27..f5ed0aa815a 100644
---
a/flink-streaming-java/src/test/java/org/apache/flink/streaming/runtime/partitioner/RebalancePartitionerTest.java
+++
b/flink-streaming-java/src/test/java/org/apache/flink/streaming/runtime/partitioner/RebalancePartitionerTest.java
@@ -18,46 +18,29 @@
package org.apache.flink.streaming.runtime.partitioner;
import org.apache.flink.api.java.tuple.Tuple;
-import org.apache.flink.runtime.plugable.SerializationDelegate;
-import org.apache.flink.streaming.runtime.streamrecord.StreamRecord;
-import org.junit.Before;
import org.junit.Test;
-import static org.junit.Assert.assertEquals;
import static org.junit.Assert.assertTrue;
/**
* Tests for {@link RebalancePartitioner}.
*/
-public class RebalancePartitionerTest {
+public class RebalancePartitionerTest extends StreamPartitionerTest {
- private RebalancePartitioner<Tuple> distributePartitioner;
- private StreamRecord<Tuple> streamRecord = new
StreamRecord<Tuple>(null);
- private SerializationDelegate<StreamRecord<Tuple>> sd = new
SerializationDelegate<StreamRecord<Tuple>>(
- null);
-
- @Before
- public void setPartitioner() {
- distributePartitioner = new RebalancePartitioner<Tuple>();
- }
-
- @Test
- public void testSelectChannelsLength() {
- sd.setInstance(streamRecord);
- assertEquals(1, distributePartitioner.selectChannels(sd,
1).length);
- assertEquals(1, distributePartitioner.selectChannels(sd,
2).length);
- assertEquals(1, distributePartitioner.selectChannels(sd,
1024).length);
+ @Override
+ public StreamPartitioner<Tuple> createPartitioner() {
+ return new RebalancePartitioner<>();
}
@Test
public void testSelectChannelsInterval() {
- sd.setInstance(streamRecord);
- int initialChannel = distributePartitioner.selectChannels(sd,
3)[0];
+ int initialChannel =
streamPartitioner.selectChannels(serializationDelegate, 3)[0];
assertTrue(0 <= initialChannel);
assertTrue(3 > initialChannel);
- assertEquals((initialChannel + 1) % 3,
distributePartitioner.selectChannels(sd, 3)[0]);
- assertEquals((initialChannel + 2) % 3,
distributePartitioner.selectChannels(sd, 3)[0]);
- assertEquals((initialChannel + 3) % 3,
distributePartitioner.selectChannels(sd, 3)[0]);
+
+ assertSelectedChannel((initialChannel + 1) % 3, 3);
+ assertSelectedChannel((initialChannel + 2) % 3, 3);
+ assertSelectedChannel((initialChannel + 3) % 3, 3);
}
}
diff --git
a/flink-streaming-java/src/test/java/org/apache/flink/streaming/runtime/partitioner/RescalePartitionerTest.java
b/flink-streaming-java/src/test/java/org/apache/flink/streaming/runtime/partitioner/RescalePartitionerTest.java
index 01d88ef1a87..212ffbd335c 100644
---
a/flink-streaming-java/src/test/java/org/apache/flink/streaming/runtime/partitioner/RescalePartitionerTest.java
+++
b/flink-streaming-java/src/test/java/org/apache/flink/streaming/runtime/partitioner/RescalePartitionerTest.java
@@ -36,16 +36,12 @@
import org.apache.flink.runtime.jobgraph.JobGraph;
import org.apache.flink.runtime.jobgraph.JobVertex;
import org.apache.flink.runtime.jobmanager.scheduler.Scheduler;
-import org.apache.flink.runtime.plugable.SerializationDelegate;
import org.apache.flink.runtime.testingUtils.TestingUtils;
import org.apache.flink.streaming.api.datastream.DataStream;
import org.apache.flink.streaming.api.environment.StreamExecutionEnvironment;
import org.apache.flink.streaming.api.functions.source.ParallelSourceFunction;
-import org.apache.flink.streaming.runtime.streamrecord.StreamRecord;
import org.apache.flink.util.Collector;
-import org.apache.flink.util.TestLogger;
-import org.junit.Before;
import org.junit.Test;
import java.util.HashMap;
@@ -62,33 +58,19 @@
* Tests for {@link RescalePartitioner}.
*/
@SuppressWarnings("serial")
-public class RescalePartitionerTest extends TestLogger {
+public class RescalePartitionerTest extends StreamPartitionerTest {
- private RescalePartitioner<Tuple> distributePartitioner;
- private StreamRecord<Tuple> streamRecord = new
StreamRecord<Tuple>(null);
- private SerializationDelegate<StreamRecord<Tuple>> sd = new
SerializationDelegate<StreamRecord<Tuple>>(
- null);
-
- @Before
- public void setPartitioner() {
- distributePartitioner = new RescalePartitioner<Tuple>();
- }
-
- @Test
- public void testSelectChannelsLength() {
- sd.setInstance(streamRecord);
- assertEquals(1, distributePartitioner.selectChannels(sd,
1).length);
- assertEquals(1, distributePartitioner.selectChannels(sd,
2).length);
- assertEquals(1, distributePartitioner.selectChannels(sd,
1024).length);
+ @Override
+ public StreamPartitioner<Tuple> createPartitioner() {
+ return new RescalePartitioner<>();
}
@Test
public void testSelectChannelsInterval() {
- sd.setInstance(streamRecord);
- assertEquals(0, distributePartitioner.selectChannels(sd, 3)[0]);
- assertEquals(1, distributePartitioner.selectChannels(sd, 3)[0]);
- assertEquals(2, distributePartitioner.selectChannels(sd, 3)[0]);
- assertEquals(0, distributePartitioner.selectChannels(sd, 3)[0]);
+ assertSelectedChannel(0, 3);
+ assertSelectedChannel(1, 3);
+ assertSelectedChannel(2, 3);
+ assertSelectedChannel(0, 3);
}
@Test
diff --git
a/flink-streaming-java/src/test/java/org/apache/flink/streaming/runtime/partitioner/ShufflePartitionerTest.java
b/flink-streaming-java/src/test/java/org/apache/flink/streaming/runtime/partitioner/ShufflePartitionerTest.java
index 238ec4b3fa2..5198ecf7ac9 100644
---
a/flink-streaming-java/src/test/java/org/apache/flink/streaming/runtime/partitioner/ShufflePartitionerTest.java
+++
b/flink-streaming-java/src/test/java/org/apache/flink/streaming/runtime/partitioner/ShufflePartitionerTest.java
@@ -18,47 +18,33 @@
package org.apache.flink.streaming.runtime.partitioner;
import org.apache.flink.api.java.tuple.Tuple;
-import org.apache.flink.runtime.plugable.SerializationDelegate;
-import org.apache.flink.streaming.runtime.streamrecord.StreamRecord;
-import org.junit.Before;
import org.junit.Test;
-import static org.junit.Assert.assertEquals;
import static org.junit.Assert.assertTrue;
/**
* Tests for {@link ShufflePartitioner}.
*/
-public class ShufflePartitionerTest {
+public class ShufflePartitionerTest extends StreamPartitionerTest {
- private ShufflePartitioner<Tuple> shufflePartitioner;
- private StreamRecord<Tuple> streamRecord = new
StreamRecord<Tuple>(null);
- private SerializationDelegate<StreamRecord<Tuple>> sd = new
SerializationDelegate<StreamRecord<Tuple>>(
- null);
-
- @Before
- public void setPartitioner() {
- shufflePartitioner = new ShufflePartitioner<Tuple>();
- }
-
- @Test
- public void testSelectChannelsLength() {
- sd.setInstance(streamRecord);
- assertEquals(1, shufflePartitioner.selectChannels(sd,
1).length);
- assertEquals(1, shufflePartitioner.selectChannels(sd,
2).length);
- assertEquals(1, shufflePartitioner.selectChannels(sd,
1024).length);
+ @Override
+ public StreamPartitioner<Tuple> createPartitioner() {
+ return new ShufflePartitioner<>();
}
@Test
public void testSelectChannelsInterval() {
- sd.setInstance(streamRecord);
- assertEquals(0, shufflePartitioner.selectChannels(sd, 1)[0]);
+ assertSelectedChannel(0, 1);
+
+ assertTrue(0 <= selectChannel(2));
+ assertTrue(2 > selectChannel(2));
- assertTrue(0 <= shufflePartitioner.selectChannels(sd, 2)[0]);
- assertTrue(2 > shufflePartitioner.selectChannels(sd, 2)[0]);
+ assertTrue(0 <= selectChannel(1024));
+ assertTrue(1024 > selectChannel(1024));
+ }
- assertTrue(0 <= shufflePartitioner.selectChannels(sd, 1024)[0]);
- assertTrue(1024 > shufflePartitioner.selectChannels(sd,
1024)[0]);
+ private int selectChannel(int numberOfChannels) {
+ return streamPartitioner.selectChannels(serializationDelegate,
numberOfChannels)[0];
}
}
diff --git
a/flink-streaming-java/src/test/java/org/apache/flink/streaming/runtime/partitioner/StreamPartitionerTest.java
b/flink-streaming-java/src/test/java/org/apache/flink/streaming/runtime/partitioner/StreamPartitionerTest.java
new file mode 100644
index 00000000000..f98b0a524f5
--- /dev/null
+++
b/flink-streaming-java/src/test/java/org/apache/flink/streaming/runtime/partitioner/StreamPartitionerTest.java
@@ -0,0 +1,59 @@
+/*
+ * 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.runtime.partitioner;
+
+import org.apache.flink.api.java.tuple.Tuple;
+import org.apache.flink.runtime.plugable.SerializationDelegate;
+import org.apache.flink.streaming.runtime.streamrecord.StreamRecord;
+import org.apache.flink.util.TestLogger;
+
+import org.junit.Before;
+import org.junit.Test;
+
+import static org.junit.Assert.assertEquals;
+
+/**
+ * Tests for different {@link StreamPartitioner} implementations.
+ */
+public abstract class StreamPartitionerTest extends TestLogger {
+
+ protected final StreamPartitioner<Tuple> streamPartitioner =
createPartitioner();
+ protected final StreamRecord<Tuple> streamRecord = new
StreamRecord<>(null);
+ protected final SerializationDelegate<StreamRecord<Tuple>>
serializationDelegate =
+ new SerializationDelegate<>(null);
+
+ abstract StreamPartitioner<Tuple> createPartitioner();
+
+ @Before
+ public void setup() {
+ serializationDelegate.setInstance(streamRecord);
+ }
+
+ @Test
+ public void testSelectChannelsLength() {
+ assertEquals(1,
streamPartitioner.selectChannels(serializationDelegate, 1).length);
+ assertEquals(1,
streamPartitioner.selectChannels(serializationDelegate, 2).length);
+ assertEquals(1,
streamPartitioner.selectChannels(serializationDelegate, 1024).length);
+ }
+
+ protected void assertSelectedChannel(int expectedChannel, int
numberOfChannels) {
+ int[] actualResult =
streamPartitioner.selectChannels(serializationDelegate, numberOfChannels);
+ assertEquals(1, actualResult.length);
+ assertEquals(expectedChannel, actualResult[0]);
+ }
+}
----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on GitHub and use the
URL above to go to the specific comment.
For queries about this service, please contact Infrastructure at:
[email protected]
> Abstract the StreamPartitionerTest for common codes
> ---------------------------------------------------
>
> Key: FLINK-10800
> URL: https://issues.apache.org/jira/browse/FLINK-10800
> Project: Flink
> Issue Type: Sub-task
> Components: Network, Tests
> Affects Versions: 1.8.0
> Reporter: zhijiang
> Assignee: zhijiang
> Priority: Minor
> Labels: pull-request-available
>
> There are many duplicated codes in current {{ForwardPartitionerTest}},
> {{RescalePartitionerTest}}, {{RebalancePartitionerTest}},
> {{RebalancePartitionerTest}}, etc.
> The {{StreamPartitionerTest}} is defined for abstracting the common codes,
> this is the preparation work for the FLINK-10790.
--
This message was sent by Atlassian JIRA
(v7.6.3#76005)