Skip to content

Commit 8c76523

Browse files
committed
[Python] Deprecate --enable_streaming_engine flag since it is always enabled. Remove weird verification if experiment is set.
1 parent 4da46b0 commit 8c76523

File tree

3 files changed

+5
-21
lines changed

3 files changed

+5
-21
lines changed

sdks/python/apache_beam/options/pipeline_options.py

Lines changed: 2 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -1021,9 +1021,9 @@ def _add_argparse_args(cls, parser):
10211021
'updating-a-pipeline')
10221022
parser.add_argument(
10231023
'--enable_streaming_engine',
1024-
default=False,
1024+
default=True,
10251025
action='store_true',
1026-
help='Enable Windmill Service for this Dataflow job. ')
1026+
help='Deprecated. All Python pipelines use Streaming Engine. ')
10271027
parser.add_argument(
10281028
'--dataflow_kms_key',
10291029
default=None,

sdks/python/apache_beam/runners/dataflow/dataflow_runner.py

Lines changed: 2 additions & 17 deletions
Original file line numberDiff line numberDiff line change
@@ -633,23 +633,8 @@ def _check_and_add_missing_streaming_options(options):
633633
# Runner v2 only supports using streaming engine (aka windmill service)
634634
if options.view_as(StandardOptions).streaming:
635635
debug_options = options.view_as(DebugOptions)
636-
google_cloud_options = options.view_as(GoogleCloudOptions)
637-
if (not google_cloud_options.enable_streaming_engine and
638-
(debug_options.lookup_experiment("enable_windmill_service") or
639-
debug_options.lookup_experiment("enable_streaming_engine"))):
640-
raise ValueError(
641-
"""Streaming engine both disabled and enabled:
642-
--enable_streaming_engine flag is not set, but
643-
enable_windmill_service and/or enable_streaming_engine experiments
644-
are present. It is recommended you only set the
645-
--enable_streaming_engine flag.""")
646-
647-
# Ensure that if we detected a streaming pipeline that streaming specific
648-
# options and experiments.
649-
options.view_as(StandardOptions).streaming = True
650-
google_cloud_options.enable_streaming_engine = True
651-
debug_options.add_experiment("enable_streaming_engine")
652-
debug_options.add_experiment("enable_windmill_service")
636+
debug_options.add_experiment('enable_streaming_engine')
637+
debug_options.add_experiment('enable_windmill_service')
653638

654639

655640
def _is_runner_v2_disabled(options):

sdks/python/apache_beam/runners/dataflow/dataflow_runner_test.py

Lines changed: 1 addition & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -421,10 +421,9 @@ def test_min_cpu_platform_flag_is_propagated_to_experiments(self):
421421
'min_cpu_platform=Intel Haswell',
422422
remote_runner.job.options.view_as(DebugOptions).experiments)
423423

424-
def test_streaming_engine_flag_adds_windmill_experiments(self):
424+
def test_streaming_adds_windmill_experiments(self):
425425
remote_runner = DataflowRunner()
426426
self.default_properties.append('--streaming')
427-
self.default_properties.append('--enable_streaming_engine')
428427
self.default_properties.append('--experiment=some_other_experiment')
429428

430429
with Pipeline(remote_runner, PipelineOptions(self.default_properties)) as p:

0 commit comments

Comments
 (0)