You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@beam.apache.org by tv...@apache.org on 2022/08/17 17:07:25 UTC

[beam] branch master updated: Fix direct running mode multi_processing on win32 (#22730)

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

tvalentyn pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/beam.git


The following commit(s) were added to refs/heads/master by this push:
     new 3602c299516 Fix direct running mode multi_processing on win32 (#22730)
3602c299516 is described below

commit 3602c299516da0f3ca3848ed1370e7cc0ffae81a
Author: Yi Hu <ya...@google.com>
AuthorDate: Wed Aug 17 13:07:18 2022 -0400

    Fix direct running mode multi_processing on win32 (#22730)
---
 sdks/python/apache_beam/runners/portability/local_job_service.py    | 5 ++++-
 sdks/python/apache_beam/runners/portability/portable_runner_test.py | 4 ----
 2 files changed, 4 insertions(+), 5 deletions(-)

diff --git a/sdks/python/apache_beam/runners/portability/local_job_service.py b/sdks/python/apache_beam/runners/portability/local_job_service.py
index 84d1ded714d..86d14c771d5 100644
--- a/sdks/python/apache_beam/runners/portability/local_job_service.py
+++ b/sdks/python/apache_beam/runners/portability/local_job_service.py
@@ -184,7 +184,10 @@ class SubprocessSdkWorker(object):
       control_address,
       provision_info,
       worker_id=None):
-    self._worker_command_line = worker_command_line
+    # worker_command_line is of bytes type received from grpc. It was encoded in
+    # apache_beam.transforms.environments.SubprocessSDKEnvironment earlier.
+    # decode it back as subprocess.Popen does not support bytes args in win32.
+    self._worker_command_line = worker_command_line.decode('utf-8')
     self._control_address = control_address
     self._provision_info = provision_info
     self._worker_id = worker_id
diff --git a/sdks/python/apache_beam/runners/portability/portable_runner_test.py b/sdks/python/apache_beam/runners/portability/portable_runner_test.py
index 78a603eef60..1ee42418cda 100644
--- a/sdks/python/apache_beam/runners/portability/portable_runner_test.py
+++ b/sdks/python/apache_beam/runners/portability/portable_runner_test.py
@@ -25,7 +25,6 @@ import time
 import unittest
 
 import grpc
-import pytest
 
 import apache_beam as beam
 from apache_beam.options.pipeline_options import DebugOptions
@@ -263,9 +262,6 @@ class PortableRunnerTestWithExternalEnv(PortableRunnerTest):
     return options
 
 
-@pytest.mark.skipif(
-    sys.platform == "win32",
-    reason="[https://github.com/apache/beam/issues/20427]")
 class PortableRunnerTestWithSubprocesses(PortableRunnerTest):
   _use_subprocesses = True