diff --git a/sdks/python/apache_beam/runners/portability/flink_runner_test.py b/sdks/python/apache_beam/runners/portability/flink_runner_test.py index a34bb571f452..f038cf678392 100644 --- a/sdks/python/apache_beam/runners/portability/flink_runner_test.py +++ b/sdks/python/apache_beam/runners/portability/flink_runner_test.py @@ -202,7 +202,7 @@ def test_read(self): def test_no_subtransform_composite(self): raise unittest.SkipTest("BEAM-4781") - def test_external_transforms(self): + def test_external_transform(self): with self.create_pipeline() as p: res = ( p @@ -213,6 +213,7 @@ def test_external_transforms(self): assert_that(res, equal_to([i for i in range(1, 10)])) + def test_expand_kafka_read(self): # We expect to fail here because we do not have a Kafka cluster handy. # Nevertheless, we check that the transform is expanded by the # ExpansionService and that the pipeline fails during execution. @@ -239,6 +240,7 @@ def test_external_transforms(self): 'Expected to fail due to invalid bootstrap.servers, but ' 'failed due to:\n%s' % str(ctx.exception)) + def test_expand_kafka_write(self): # We just test the expansion but do not execute. # pylint: disable=expression-not-assigned (