You are viewing a plain text version of this content. The canonical link for it is here.
Posted to issues@flink.apache.org by aljoscha <gi...@git.apache.org> on 2018/01/26 13:11:04 UTC

[GitHub] flink pull request #5369: [FLINK-8407][DataStream]Setting the parallelism af...

Github user aljoscha commented on a diff in the pull request:

    https://github.com/apache/flink/pull/5369#discussion_r164106802
  
    --- Diff: flink-streaming-scala/src/test/scala/org/apache/flink/streaming/api/scala/DataStreamTest.scala ---
    @@ -299,6 +299,19 @@ class DataStreamTest extends AbstractTestBase {
         assert(4 == env.getStreamGraph.getStreamNode(sink.getTransformation.getId).getParallelism)
       }
     
    +  /**
    +    * Tests setting the parallelism after a partitioning operation (e.g., broadcast, rescale)
    +    * should fail.
    +    */
    +  @Test(expected = classOf[UnsupportedOperationException])
    +  def testParallelismFailAfterPartitioning(): Unit = {
    +    val env: StreamExecutionEnvironment = StreamExecutionEnvironment.getExecutionEnvironment
    +
    +    val src = env.fromElements(new Tuple2[Long, Long](0L, 0L))
    +    val map = src.map(_ => (0L, 0L))
    +    map.broadcast.setParallelism(1)
    --- End diff --
    
    I think this test could be made more fine-grained by verifying that `setParallelism()` is in fact throwing the exception. As it is now, any parts of the code could throw the exception and the test would succeed.
    
    You could use the `ExpectedException` `@Rule`, as for example here: https://github.com/apache/flink/blob/db440f2434423a23207ba666b33f4ccb55adede5/flink-runtime/src/test/java/org/apache/flink/runtime/io/network/buffer/BufferPoolFactoryTest.java#L53


---