diff --git a/sdks/python/apache_beam/testing/benchmarks/nexmark/nexmark_launcher.py b/sdks/python/apache_beam/testing/benchmarks/nexmark/nexmark_launcher.py index 89b6dce2b58f..e4babe5f42e8 100644 --- a/sdks/python/apache_beam/testing/benchmarks/nexmark/nexmark_launcher.py +++ b/sdks/python/apache_beam/testing/benchmarks/nexmark/nexmark_launcher.py @@ -69,6 +69,7 @@ from apache_beam.options.pipeline_options import PipelineOptions from apache_beam.options.pipeline_options import SetupOptions from apache_beam.options.pipeline_options import StandardOptions +from apache_beam.options.pipeline_options import TypeOptions from apache_beam.runners import PipelineState from apache_beam.testing.benchmarks.nexmark import nexmark_util from apache_beam.testing.benchmarks.nexmark.monitor import Monitor @@ -180,6 +181,7 @@ def parse_args(self): # Usage with Dataflow requires a project to be supplied. self.project = self.pipeline_options.view_as(GoogleCloudOptions).project self.streaming = self.pipeline_options.view_as(StandardOptions).streaming + self.pipeline_options.view_as(TypeOptions).allow_unsafe_triggers = True if self.streaming: if self.args.subscription_name is None or self.project is None: