Repository: beam
Updated Branches:
  refs/heads/master ad90d91cb -> 311425ced


Fail when --streaming option is used.

Currently, none of the runners for Python SDK supports streaming. Fail
at pipeline validation time if '--streaming' option is set. This is to
prevent user confusion, until streaming execution is implemented.


Project: http://git-wip-us.apache.org/repos/asf/beam/repo
Commit: http://git-wip-us.apache.org/repos/asf/beam/commit/31a5754b
Tree: http://git-wip-us.apache.org/repos/asf/beam/tree/31a5754b
Diff: http://git-wip-us.apache.org/repos/asf/beam/diff/31a5754b

Branch: refs/heads/master
Commit: 31a5754bd73a172d5a3f50adab060ebc2c53f81a
Parents: ad90d91
Author: Ahmet Altay <[email protected]>
Authored: Wed Feb 1 17:55:54 2017 -0800
Committer: Ahmet Altay <[email protected]>
Committed: Thu Feb 2 09:47:30 2017 -0800

----------------------------------------------------------------------
 sdks/python/apache_beam/utils/pipeline_options.py            | 8 ++++++++
 .../apache_beam/utils/pipeline_options_validator_test.py     | 8 ++++++++
 2 files changed, 16 insertions(+)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/beam/blob/31a5754b/sdks/python/apache_beam/utils/pipeline_options.py
----------------------------------------------------------------------
diff --git a/sdks/python/apache_beam/utils/pipeline_options.py 
b/sdks/python/apache_beam/utils/pipeline_options.py
index 16b1640..f7d7597 100644
--- a/sdks/python/apache_beam/utils/pipeline_options.py
+++ b/sdks/python/apache_beam/utils/pipeline_options.py
@@ -189,6 +189,14 @@ class StandardOptions(PipelineOptions):
                         action='store_true',
                         help='Whether to enable streaming mode.')
 
+  # TODO(BEAM-1265): Remove this error, once at least one runner supports
+  # streaming pipelines.
+  def validate(self, validator):
+    errors = []
+    if self.view_as(StandardOptions).streaming:
+      errors.append('Streaming pipelines are not supported.')
+    return errors
+
 
 class TypeOptions(PipelineOptions):
 

http://git-wip-us.apache.org/repos/asf/beam/blob/31a5754b/sdks/python/apache_beam/utils/pipeline_options_validator_test.py
----------------------------------------------------------------------
diff --git a/sdks/python/apache_beam/utils/pipeline_options_validator_test.py 
b/sdks/python/apache_beam/utils/pipeline_options_validator_test.py
index ffed048..cb7cd84 100644
--- a/sdks/python/apache_beam/utils/pipeline_options_validator_test.py
+++ b/sdks/python/apache_beam/utils/pipeline_options_validator_test.py
@@ -300,6 +300,14 @@ class SetupTest(unittest.TestCase):
     errors = validator.validate()
     self.assertFalse(errors)
 
+  def test_streaming(self):
+    pipeline_options = PipelineOptions(['--streaming'])
+    runner = MockRunners.TestDataflowRunner()
+    validator = PipelineOptionsValidator(pipeline_options, runner)
+    errors = validator.validate()
+
+    self.assertIn('Streaming pipelines are not supported.', errors)
+
   def test_test_matcher(self):
     def get_validator(matcher):
       options = ['--project=example:example',

Reply via email to