[ 
https://issues.apache.org/jira/browse/BEAM-3645?focusedWorklogId=282598&page=com.atlassian.jira.plugin.system.issuetabpanels:worklog-tabpanel#worklog-282598
 ]

ASF GitHub Bot logged work on BEAM-3645:
----------------------------------------

                Author: ASF GitHub Bot
            Created on: 25/Jul/19 10:06
            Start Date: 25/Jul/19 10:06
    Worklog Time Spent: 10m 
      Work Description: robertwb commented on pull request #8979: [BEAM-3645] 
add multiplexing for python FnApiRunner
URL: https://github.com/apache/beam/pull/8979#discussion_r307175915
 
 

 ##########
 File path: sdks/python/apache_beam/runners/portability/fn_api_runner.py
 ##########
 @@ -82,6 +82,54 @@
         beam.transforms.window.GlobalWindows.windowed_value(b''))
 
 
+class ControlConnection(object):
+
+  _uid_counter = 0
+  _lock = threading.Lock()
+
+  def __init__(self):
+    self._push_queue = queue.Queue()
+    self._input = None
+    self._input_is_set = False
+    self._futures_by_id = dict()
+    self._read_thread = threading.Thread(
+        name='beam_control_read', target=self._read)
+    self._state = BeamFnControlServicer.STARTED_STATE
 
 Review comment:
   Here's where your hanging issues might be coming from (or at the very least 
this is a risk for the future): the state shouldn't be started until the thread 
is running. 
   
   What you probably need to do is add a lock in set_input that updates 
self._state and starts the thread running (iff state is not already done), and 
then also lock in close to only stop the read thread if it's been started. 
   
   If you do this, you may be able to completely get rid of the locks in 
BeamFnControlServicer as well. Doneness is a per-connection property, not a 
per-server one. 
 
----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
[email protected]


Issue Time Tracking
-------------------

    Worklog Id:     (was: 282598)
    Time Spent: 31h 50m  (was: 31h 40m)

> Support multi-process execution on the FnApiRunner
> --------------------------------------------------
>
>                 Key: BEAM-3645
>                 URL: https://issues.apache.org/jira/browse/BEAM-3645
>             Project: Beam
>          Issue Type: Improvement
>          Components: sdk-py-core
>    Affects Versions: 2.2.0, 2.3.0
>            Reporter: Charles Chen
>            Assignee: Hannah Jiang
>            Priority: Major
>             Fix For: 2.15.0
>
>          Time Spent: 31h 50m
>  Remaining Estimate: 0h
>
> https://issues.apache.org/jira/browse/BEAM-3644 gave us a 15x performance 
> gain over the previous DirectRunner.  We can do even better in multi-core 
> environments by supporting multi-process execution in the FnApiRunner, to 
> scale past Python GIL limitations.



--
This message was sent by Atlassian JIRA
(v7.6.14#76016)

Reply via email to