[
https://issues.apache.org/jira/browse/BEAM-3645?focusedWorklogId=281084&page=com.atlassian.jira.plugin.system.issuetabpanels:worklog-tabpanel#worklog-281084
]
ASF GitHub Bot logged work on BEAM-3645:
----------------------------------------
Author: ASF GitHub Bot
Created on: 23/Jul/19 15:20
Start Date: 23/Jul/19 15:20
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_r306368474
##########
File path: sdks/python/apache_beam/runners/portability/fn_api_runner.py
##########
@@ -92,34 +140,56 @@ class
BeamFnControlServicer(beam_fn_api_pb2_grpc.BeamFnControlServicer):
_DONE_MARKER = object()
def __init__(self):
- self._push_queue = queue.Queue()
- self._futures_by_id = dict()
- self._read_thread = threading.Thread(
- name='beam_control_read', target=self._read)
+ self._lock = threading.Lock()
self._uid_counter = 0
self._state = self.UNSTARTED_STATE
- self._lock = threading.Lock()
+ # following self._req_* variables are used for debugging purpose, data is
+ # added only when self._log_req is True.
+ self._req_sent = collections.defaultdict(int)
+ self._req_worker_mapping = {}
+ self._log_req = True if logging.getLevelName(
+ logging.getLogger().getEffectiveLevel()) == 'DEBUG' else False
+ self._conn_handler = ConnectionHandler()
def Control(self, iterator, context):
with self._lock:
if self._state == self.DONE_STATE:
return
else:
self._state = self.STARTED_STATE
- self._inputs = iterator
- # Note: We only support one client for now.
- self._read_thread.start()
+
+ worker_id = dict(context.invocation_metadata()).get('worker_id')
+ if not worker_id:
+ raise RuntimeError('All workers communicate through gRPC should have '
+ 'worker_id. Received None.')
+
+ control_conn = self._conn_handler.get_conn_handler_by_id(worker_id)
+ control_conn.set_input(iterator)
+ if not control_conn.read_thread.isAlive():
+ control_conn.read_thread.start()
+
while True:
- to_push = self._push_queue.get()
+ to_push = control_conn.get_req()
if to_push is self._DONE_MARKER:
return
yield to_push
+ if self._log_req:
+ self._req_sent[to_push.instruction_id] += 1
+
+ def _dispatch(self, item, dest_worker_id, future):
Review comment:
You should be able to get rid of `BeamFnControlServicer.{push,_dispatch}`.
Instead, to push a message, users would call
`control_servicer.get_connection(worker_id).push(request)`. Most of the time
they wouldn't actually even bother holding a handle to the control_servicer,
rather they'd just hold a ControlConnection and do
`self._control_connection.push(...)`.
----------------------------------------------------------------
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: 281084)
Time Spent: 27h 40m (was: 27.5h)
> 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: 27h 40m
> 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)