This is an automated email from the ASF dual-hosted git repository.

robertwb pushed a commit to branch release-2.29.0
in repository https://gitbox.apache.org/repos/asf/beam.git


The following commit(s) were added to refs/heads/release-2.29.0 by this push:
     new cbcf5ac  [BEAM-11932] Rename service_options to 
dataflow_service_options.
     new 35010c1  Merge pull request #14461 from [BEAM-11932] Rename 
service_options to dataflow_service_options.
cbcf5ac is described below

commit cbcf5ac3a44d7f74323b8b7aed5b2016e30f09ca
Author: Robert Bradshaw <[email protected]>
AuthorDate: Fri Apr 2 10:13:50 2021 -0700

    [BEAM-11932] Rename service_options to dataflow_service_options.
---
 .../dataflow/DataflowPipelineTranslator.java       |  2 +-
 .../dataflow/options/DataflowPipelineOptions.java  |  4 ++--
 .../dataflow/DataflowPipelineTranslatorTest.java   |  8 +++----
 .../python/apache_beam/options/pipeline_options.py |  6 +++---
 .../apache_beam/options/pipeline_options_test.py   | 25 +++++++++++++++-------
 .../runners/dataflow/internal/apiclient.py         |  4 ++--
 .../runners/dataflow/internal/apiclient_test.py    |  4 ++--
 7 files changed, 31 insertions(+), 22 deletions(-)

diff --git 
a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/DataflowPipelineTranslator.java
 
b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/DataflowPipelineTranslator.java
index 0fd8d80..c096692 100644
--- 
a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/DataflowPipelineTranslator.java
+++ 
b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/DataflowPipelineTranslator.java
@@ -334,7 +334,7 @@ public class DataflowPipelineTranslator {
 
       Environment environment = new Environment();
       job.setEnvironment(environment);
-      job.getEnvironment().setServiceOptions(options.getServiceOptions());
+      
job.getEnvironment().setServiceOptions(options.getDataflowServiceOptions());
 
       WorkerPool workerPool = new WorkerPool();
 
diff --git 
a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/options/DataflowPipelineOptions.java
 
b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/options/DataflowPipelineOptions.java
index 8434ff7..7d3be45 100644
--- 
a/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/options/DataflowPipelineOptions.java
+++ 
b/runners/google-cloud-dataflow-java/src/main/java/org/apache/beam/runners/dataflow/options/DataflowPipelineOptions.java
@@ -115,9 +115,9 @@ public interface DataflowPipelineOptions
   @Description(
       "Service options are set by the user and configure the service. This "
           + "decouples service side feature availability from the Apache Beam 
release cycle.")
-  List<String> getServiceOptions();
+  List<String> getDataflowServiceOptions();
 
-  void setServiceOptions(List<String> options);
+  void setDataflowServiceOptions(List<String> options);
 
   /** Run the job as a specific service account, instead of the default GCE 
robot. */
   @Hidden
diff --git 
a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/DataflowPipelineTranslatorTest.java
 
b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/DataflowPipelineTranslatorTest.java
index 8015ab3..b296f6e 100644
--- 
a/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/DataflowPipelineTranslatorTest.java
+++ 
b/runners/google-cloud-dataflow-java/src/test/java/org/apache/beam/runners/dataflow/DataflowPipelineTranslatorTest.java
@@ -1293,12 +1293,12 @@ public class DataflowPipelineTranslatorTest implements 
Serializable {
   }
 
   @Test
-  public void testServiceOptionsSet() throws IOException {
-    final List<String> serviceOptions =
+  public void testDataflowServiceOptionsSet() throws IOException {
+    final List<String> dataflowServiceOptions =
         Stream.of("whizz=bang", "foo=bar").collect(Collectors.toList());
 
     DataflowPipelineOptions options = buildPipelineOptions();
-    options.setServiceOptions(serviceOptions);
+    options.setDataflowServiceOptions(dataflowServiceOptions);
 
     Pipeline p = buildPipeline(options);
     p.traverseTopologically(new RecordingPipelineVisitor());
@@ -1314,7 +1314,7 @@ public class DataflowPipelineTranslatorTest implements 
Serializable {
                 Collections.emptyList())
             .getJob();
 
-    assertEquals(serviceOptions, job.getEnvironment().getServiceOptions());
+    assertEquals(dataflowServiceOptions, 
job.getEnvironment().getServiceOptions());
   }
 
   private static void assertAllStepOutputsHaveUniqueIds(Job job) throws 
Exception {
diff --git a/sdks/python/apache_beam/options/pipeline_options.py 
b/sdks/python/apache_beam/options/pipeline_options.py
index 181b536..ebc5f40 100644
--- a/sdks/python/apache_beam/options/pipeline_options.py
+++ b/sdks/python/apache_beam/options/pipeline_options.py
@@ -682,9 +682,9 @@ class GoogleCloudOptions(PipelineOptions):
         choices=['COST_OPTIMIZED', 'SPEED_OPTIMIZED'],
         help='Set the Flexible Resource Scheduling mode')
     parser.add_argument(
-        '--service_option',
-        '--service_options',
-        dest='service_options',
+        '--dataflow_service_option',
+        '--dataflow_service_options',
+        dest='dataflow_service_options',
         action='append',
         default=None,
         help=(
diff --git a/sdks/python/apache_beam/options/pipeline_options_test.py 
b/sdks/python/apache_beam/options/pipeline_options_test.py
index ba4c951..08c4145 100644
--- a/sdks/python/apache_beam/options/pipeline_options_test.py
+++ b/sdks/python/apache_beam/options/pipeline_options_test.py
@@ -623,21 +623,30 @@ class PipelineOptionsTest(unittest.TestCase):
     mapping = options.view_as(GoogleCloudOptions).transform_name_mapping
     self.assertEqual(mapping['from'], 'to')
 
-  def test_service_options(self):
-    options = PipelineOptions(
-        ['--service_option', 'whizz=bang', '--service_option', 'beep=boop'])
+  def test_dataflow_service_options(self):
+    options = PipelineOptions([
+        '--dataflow_service_option',
+        'whizz=bang',
+        '--dataflow_service_option',
+        'beep=boop'
+    ])
     self.assertEqual(
-        sorted(options.get_all_options()['service_options']),
+        sorted(options.get_all_options()['dataflow_service_options']),
         ['beep=boop', 'whizz=bang'])
 
-    options = PipelineOptions(
-        ['--service_options', 'whizz=bang', '--service_options', 'beep=boop'])
+    options = PipelineOptions([
+        '--dataflow_service_options',
+        'whizz=bang',
+        '--dataflow_service_options',
+        'beep=boop'
+    ])
     self.assertEqual(
-        sorted(options.get_all_options()['service_options']),
+        sorted(options.get_all_options()['dataflow_service_options']),
         ['beep=boop', 'whizz=bang'])
 
     options = PipelineOptions(flags=[''])
-    self.assertEqual(options.get_all_options()['service_options'], None)
+    self.assertEqual(
+        options.get_all_options()['dataflow_service_options'], None)
 
 
 if __name__ == '__main__':
diff --git a/sdks/python/apache_beam/runners/dataflow/internal/apiclient.py 
b/sdks/python/apache_beam/runners/dataflow/internal/apiclient.py
index 6362b8c..275d68d 100644
--- a/sdks/python/apache_beam/runners/dataflow/internal/apiclient.py
+++ b/sdks/python/apache_beam/runners/dataflow/internal/apiclient.py
@@ -347,8 +347,8 @@ class Environment(object):
           dataflow.Environment.SdkPipelineOptionsValue.AdditionalProperty(
               key='display_data', value=to_json_value(items)))
 
-    if self.google_cloud_options.service_options:
-      for option in self.google_cloud_options.service_options:
+    if self.google_cloud_options.dataflow_service_options:
+      for option in self.google_cloud_options.dataflow_service_options:
         self.proto.serviceOptions.append(option)
 
   def _get_environments_from_tranforms(self):
diff --git 
a/sdks/python/apache_beam/runners/dataflow/internal/apiclient_test.py 
b/sdks/python/apache_beam/runners/dataflow/internal/apiclient_test.py
index de64be5..52a3d5f 100644
--- a/sdks/python/apache_beam/runners/dataflow/internal/apiclient_test.py
+++ b/sdks/python/apache_beam/runners/dataflow/internal/apiclient_test.py
@@ -1186,9 +1186,9 @@ class UtilTest(unittest.TestCase):
          ('/tmp/bar2', 'bar2')],
         staging_location='gs://test-location/staging')
 
-  def test_set_service_option(self):
+  def test_set_dataflow_service_option(self):
     pipeline_options = PipelineOptions([
-        '--service_option',
+        '--dataflow_service_option',
         'whizz=bang',
         '--temp_location',
         'gs://any-location/temp'

Reply via email to