Get rid of proxy process in zmq
As far as we use redis as a name service we don't need a proxy, becase we can pass binded port over name service too. Change-Id: I59bbe2b34dcedfeef113ef06d6a988e1c413405echanges/07/205007/10
parent
e2c3e36d75
commit
315e56ae2b
@ -1 +0,0 @@
|
||||
|
@ -1,44 +0,0 @@
|
||||
#!/usr/bin/env python
|
||||
|
||||
# Copyright 2011 OpenStack Foundation
|
||||
#
|
||||
# Licensed under the Apache License, Version 2.0 (the "License"); you may
|
||||
# not use this file except in compliance with the License. You may obtain
|
||||
# a copy of the License at
|
||||
#
|
||||
# http://www.apache.org/licenses/LICENSE-2.0
|
||||
#
|
||||
# Unless required by applicable law or agreed to in writing, software
|
||||
# distributed under the License is distributed on an "AS IS" BASIS, WITHOUT
|
||||
# WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. See the
|
||||
# License for the specific language governing permissions and limitations
|
||||
# under the License.
|
||||
|
||||
import contextlib
|
||||
import logging
|
||||
import sys
|
||||
|
||||
from oslo_config import cfg
|
||||
|
||||
from oslo_messaging._drivers import impl_zmq
|
||||
from oslo_messaging._drivers.zmq_driver.broker import zmq_broker
|
||||
from oslo_messaging._executors import base # FIXME(markmc)
|
||||
|
||||
CONF = cfg.CONF
|
||||
CONF.register_opts(impl_zmq.zmq_opts)
|
||||
CONF.register_opts(base._pool_opts)
|
||||
# TODO(ozamiatin): Move this option assignment to an external config file
|
||||
# Use efficient zmq poller in real-world deployment
|
||||
CONF.rpc_zmq_native = True
|
||||
|
||||
|
||||
def main():
|
||||
CONF(sys.argv[1:], project='oslo')
|
||||
logging.basicConfig(level=logging.DEBUG)
|
||||
|
||||
with contextlib.closing(zmq_broker.ZmqBroker(CONF)) as reactor:
|
||||
reactor.start()
|
||||
reactor.wait()
|
||||
|
||||
if __name__ == "__main__":
|
||||
main()
|
@ -1 +0,0 @@
|
||||
__author__ = 'ozamiatin'
|
@ -1,163 +0,0 @@
|
||||
# Copyright 2015 Mirantis, Inc.
|
||||
#
|
||||
# Licensed under the Apache License, Version 2.0 (the "License"); you may
|
||||
# not use this file except in compliance with the License. You may obtain
|
||||
# a copy of the License at
|
||||
#
|
||||
# http://www.apache.org/licenses/LICENSE-2.0
|
||||
#
|
||||
# Unless required by applicable law or agreed to in writing, software
|
||||
# distributed under the License is distributed on an "AS IS" BASIS, WITHOUT
|
||||
# WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. See the
|
||||
# License for the specific language governing permissions and limitations
|
||||
# under the License.
|
||||
|
||||
import abc
|
||||
import logging
|
||||
|
||||
import six
|
||||
|
||||
from oslo_messaging._drivers.common import RPCException
|
||||
from oslo_messaging._drivers.zmq_driver import zmq_async
|
||||
from oslo_messaging._drivers.zmq_driver import zmq_target
|
||||
from oslo_messaging._i18n import _LE, _LI
|
||||
|
||||
LOG = logging.getLogger(__name__)
|
||||
|
||||
zmq = zmq_async.import_zmq()
|
||||
|
||||
|
||||
@six.add_metaclass(abc.ABCMeta)
|
||||
class BaseProxy(object):
|
||||
|
||||
"""Base TCP-proxy.
|
||||
|
||||
TCP-proxy redirects messages received by TCP from clients to servers
|
||||
over IPC. Consists of TCP-frontend and IPC-backend objects. Runs
|
||||
in async executor.
|
||||
"""
|
||||
|
||||
def __init__(self, conf, context):
|
||||
super(BaseProxy, self).__init__()
|
||||
self.conf = conf
|
||||
self.context = context
|
||||
self.executor = zmq_async.get_executor(
|
||||
self.run, native_zmq=conf.rpc_zmq_native)
|
||||
|
||||
@abc.abstractmethod
|
||||
def run(self):
|
||||
"""Main execution point of the proxy"""
|
||||
|
||||
def start(self):
|
||||
self.executor.execute()
|
||||
|
||||
def stop(self):
|
||||
self.executor.stop()
|
||||
|
||||
def wait(self):
|
||||
self.executor.wait()
|
||||
|
||||
|
||||
@six.add_metaclass(abc.ABCMeta)
|
||||
class BaseTcpFrontend(object):
|
||||
|
||||
"""Base frontend clause.
|
||||
|
||||
TCP-frontend is a part of TCP-proxy which receives incoming
|
||||
messages from clients.
|
||||
"""
|
||||
|
||||
def __init__(self, conf, poller, context,
|
||||
socket_type=None,
|
||||
port_number=None,
|
||||
receive_meth=None):
|
||||
|
||||
"""Construct a TCP-frontend.
|
||||
|
||||
Its attributes are:
|
||||
|
||||
:param conf: Driver configuration object.
|
||||
:type conf: ConfigOpts
|
||||
:param poller: Messages poller-object green or threading.
|
||||
:type poller: ZmqPoller
|
||||
:param context: ZeroMQ context object.
|
||||
:type context: zmq.Context
|
||||
:param socket_type: ZeroMQ socket type.
|
||||
:type socket_type: int
|
||||
:param port_number: Current messaging pipeline port.
|
||||
:type port_number: int
|
||||
:param receive_meth: Receive method for poller.
|
||||
:type receive_meth: method
|
||||
"""
|
||||
|
||||
self.conf = conf
|
||||
self.poller = poller
|
||||
self.context = context
|
||||
try:
|
||||
self.frontend = self.context.socket(socket_type)
|
||||
bind_address = zmq_target.get_tcp_bind_address(port_number)
|
||||
LOG.info(_LI("Binding to TCP %s") % bind_address)
|
||||
self.frontend.bind(bind_address)
|
||||
self.poller.register(self.frontend, receive_meth)
|
||||
except zmq.ZMQError as e:
|
||||
errmsg = _LE("Could not create ZeroMQ receiver daemon. "
|
||||
"Socket may already be in use: %s") % str(e)
|
||||
LOG.error(errmsg)
|
||||
raise RPCException(errmsg)
|
||||
|
||||
def receive_incoming(self):
|
||||
message, socket = self.poller.poll(1)
|
||||
LOG.info(_LI("Message %s received."), message)
|
||||
return message
|
||||
|
||||
def close(self):
|
||||
self.frontend.close()
|
||||
|
||||
|
||||
@six.add_metaclass(abc.ABCMeta)
|
||||
class BaseBackendMatcher(object):
|
||||
|
||||
def __init__(self, conf, poller, context):
|
||||
self.conf = conf
|
||||
self.context = context
|
||||
self.backends = {}
|
||||
self.poller = poller
|
||||
|
||||
@abc.abstractmethod
|
||||
def redirect_to_backend(self, message):
|
||||
"""Redirect message"""
|
||||
|
||||
def close(self):
|
||||
if self.backends:
|
||||
for backend in self.backends.values():
|
||||
backend.close()
|
||||
|
||||
|
||||
@six.add_metaclass(abc.ABCMeta)
|
||||
class DirectBackendMatcher(BaseBackendMatcher):
|
||||
|
||||
def redirect_to_backend(self, message):
|
||||
backend, target = self._match_backend(message)
|
||||
self._send_message(backend, message, target)
|
||||
|
||||
def _match_backend(self, message):
|
||||
target = self._get_target(message)
|
||||
ipc_address = self._get_ipc_address(target)
|
||||
backend = self._create_backend(ipc_address)
|
||||
return backend, target
|
||||
|
||||
@abc.abstractmethod
|
||||
def _get_target(self, message):
|
||||
"""Extract topic from message"""
|
||||
|
||||
@abc.abstractmethod
|
||||
def _get_ipc_address(self, target):
|
||||
"""Get ipc backend address from topic"""
|
||||
|
||||
@abc.abstractmethod
|
||||
def _send_message(self, backend, message, target):
|
||||
"""Backend specific sending logic"""
|
||||
|
||||
@abc.abstractmethod
|
||||
def _create_backend(self, ipc_address):
|
||||
"""Backend specific socket opening logic"""
|
@ -1,71 +0,0 @@
|
||||
# Copyright 2015 Mirantis, Inc.
|
||||
#
|
||||
# Licensed under the Apache License, Version 2.0 (the "License"); you may
|
||||
# not use this file except in compliance with the License. You may obtain
|
||||
# a copy of the License at
|
||||
#
|
||||
# http://www.apache.org/licenses/LICENSE-2.0
|
||||
#
|
||||
# Unless required by applicable law or agreed to in writing, software
|
||||
# distributed under the License is distributed on an "AS IS" BASIS, WITHOUT
|
||||
# WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. See the
|
||||
# License for the specific language governing permissions and limitations
|
||||
# under the License.
|
||||
|
||||
import logging
|
||||
import os
|
||||
|
||||
from oslo_utils import excutils
|
||||
|
||||
from oslo_messaging._drivers.zmq_driver.broker import zmq_universal_proxy
|
||||
from oslo_messaging._drivers.zmq_driver import zmq_async
|
||||
from oslo_messaging._i18n import _LE, _LI
|
||||
|
||||
|
||||
LOG = logging.getLogger(__name__)
|
||||
|
||||
|
||||
class ZmqBroker(object):
|
||||
"""Local messaging IPC broker (nodes are still peers).
|
||||
|
||||
The main purpose is to have one TCP connection
|
||||
(one TCP port assigned for ZMQ messaging) per node.
|
||||
There could be a number of services running on a node.
|
||||
Without such broker a number of opened TCP ports used for
|
||||
messaging become unpredictable for the engine.
|
||||
|
||||
All messages are coming to TCP ROUTER socket and then
|
||||
distributed between their targets by topic via IPC.
|
||||
"""
|
||||
|
||||
def __init__(self, conf):
|
||||
super(ZmqBroker, self).__init__()
|
||||
zmq = zmq_async.import_zmq(native_zmq=conf.rpc_zmq_native)
|
||||
self.conf = conf
|
||||
self.context = zmq.Context()
|
||||
proxy = zmq_universal_proxy.UniversalProxy(conf, self.context)
|
||||
self.proxies = [proxy]
|
||||
self._create_ipc_dirs()
|
||||
|
||||
def _create_ipc_dirs(self):
|
||||
ipc_dir = self.conf.rpc_zmq_ipc_dir
|
||||
try:
|
||||
os.makedirs("%s/fanout" % ipc_dir)
|
||||
except os.error:
|
||||
if not os.path.isdir(ipc_dir):
|
||||
with excutils.save_and_reraise_exception():
|
||||
LOG.error(_LE("Required IPC directory does not exist at"
|
||||
" %s"), ipc_dir)
|
||||
|
||||
def start(self):
|
||||
for proxy in self.proxies:
|
||||
proxy.start()
|
||||
|
||||
def wait(self):
|
||||
for proxy in self.proxies:
|
||||
proxy.wait()
|
||||
|
||||
def close(self):
|
||||
LOG.info(_LI("Broker shutting down ..."))
|
||||
for proxy in self.proxies:
|
||||
proxy.stop()
|
@ -1,106 +0,0 @@
|
||||
# Copyright 2015 Mirantis, Inc.
|
||||
#
|
||||
# Licensed under the Apache License, Version 2.0 (the "License"); you may
|
||||
# not use this file except in compliance with the License. You may obtain
|
||||
# a copy of the License at
|
||||
#
|
||||
# http://www.apache.org/licenses/LICENSE-2.0
|
||||
#
|
||||
# Unless required by applicable law or agreed to in writing, software
|
||||
# distributed under the License is distributed on an "AS IS" BASIS, WITHOUT
|
||||
# WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. See the
|
||||
# License for the specific language governing permissions and limitations
|
||||
# under the License.
|
||||
|
||||
import logging
|
||||
|
||||
from oslo_messaging._drivers.common import RPCException
|
||||
import oslo_messaging._drivers.zmq_driver.broker.zmq_base_proxy as base_proxy
|
||||
from oslo_messaging._drivers.zmq_driver import zmq_async
|
||||
from oslo_messaging._drivers.zmq_driver import zmq_serializer
|
||||
from oslo_messaging._drivers.zmq_driver import zmq_target
|
||||
from oslo_messaging._i18n import _LE, _LI
|
||||
|
||||
LOG = logging.getLogger(__name__)
|
||||
|
||||
zmq = zmq_async.import_zmq()
|
||||
|
||||
|
||||
class DealerBackend(base_proxy.DirectBackendMatcher):
|
||||
|
||||
def __init__(self, conf, context, poller=None):
|
||||
if poller is None:
|
||||
poller = zmq_async.get_poller(
|
||||
native_zmq=conf.rpc_zmq_native)
|
||||
super(DealerBackend, self).__init__(conf, poller, context)
|
||||
|
||||
def _get_target(self, message):
|
||||
return zmq_serializer.get_target_from_call_message(message)
|
||||
|
||||
def _get_ipc_address(self, target):
|
||||
return zmq_target.get_ipc_address_call(self.conf, target)
|
||||
|
||||
def _send_message(self, backend, message, topic):
|
||||
# Empty needed for awaiting REP socket to work properly
|
||||
# (DEALER-REP usage specific)
|
||||
backend.send(b'', zmq.SNDMORE)
|
||||
backend.send(message.pop(0), zmq.SNDMORE)
|
||||
backend.send_string(message.pop(0), zmq.SNDMORE)
|
||||
message.pop(0) # Drop target unneeded any more
|
||||
backend.send_multipart(message)
|
||||
|
||||
def _create_backend(self, ipc_address):
|
||||
if ipc_address in self.backends:
|
||||
return self.backends[ipc_address]
|
||||
backend = self.context.socket(zmq.DEALER)
|
||||
backend.connect(ipc_address)
|
||||
self.poller.register(backend)
|
||||
self.backends[ipc_address] = backend
|
||||
return backend
|
||||
|
||||
|
||||
class FrontendTcpRouter(base_proxy.BaseTcpFrontend):
|
||||
|
||||
def __init__(self, conf, context, poller=None):
|
||||
if poller is None:
|
||||
poller = zmq_async.get_poller(
|
||||
native_zmq=conf.rpc_zmq_native)
|
||||
super(FrontendTcpRouter, self).__init__(
|
||||
conf, poller, context,
|
||||
socket_type=zmq.ROUTER,
|
||||
port_number=conf.rpc_zmq_port,
|
||||
receive_meth=self._receive_message)
|
||||
|
||||
def _receive_message(self, socket):
|
||||
|
||||
try:
|
||||
reply_id = socket.recv()
|
||||
empty = socket.recv()
|
||||
assert empty == b'', "Empty delimiter expected"
|
||||
msg_type = socket.recv_string()
|
||||
target_dict = socket.recv_json()
|
||||
target = zmq_target.target_from_dict(target_dict)
|
||||
other = socket.recv_multipart()
|
||||
except zmq.ZMQError as e:
|
||||
LOG.error(_LE("Error receiving message %s") % str(e))
|
||||
return None
|
||||
|
||||
if msg_type == zmq_serializer.FANOUT_TYPE:
|
||||
other.insert(0, zmq_target.target_to_str(target).encode("utf-8"))
|
||||
|
||||
return [reply_id, msg_type, target] + other
|
||||
|
||||
@staticmethod
|
||||
def _reduce_empty(reply):
|
||||
reply.pop(0)
|
||||
return reply
|
||||
|
||||
def redirect_outgoing_reply(self, reply):
|
||||
self._reduce_empty(reply)
|
||||
try:
|
||||
self.frontend.send_multipart(reply)
|
||||
LOG.info(_LI("Redirecting reply to client %s") % reply)
|
||||
except zmq.ZMQError:
|
||||
errmsg = _LE("Failed redirecting reply to client %s") % reply
|
||||
LOG.error(errmsg)
|
||||
raise RPCException(errmsg)
|
@ -1,38 +0,0 @@
|
||||
# Copyright 2015 Mirantis, Inc.
|
||||
#
|
||||
# Licensed under the Apache License, Version 2.0 (the "License"); you may
|
||||
# not use this file except in compliance with the License. You may obtain
|
||||
# a copy of the License at
|
||||
#
|
||||
# http://www.apache.org/licenses/LICENSE-2.0
|
||||
#
|
||||
# Unless required by applicable law or agreed to in writing, software
|
||||
# distributed under the License is distributed on an "AS IS" BASIS, WITHOUT
|
||||
# WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. See the
|
||||
# License for the specific language governing permissions and limitations
|
||||
# under the License.
|
||||
|
||||
|
||||
import oslo_messaging._drivers.zmq_driver.broker.zmq_base_proxy as base_proxy
|
||||
from oslo_messaging._drivers.zmq_driver import zmq_async
|
||||
from oslo_messaging._drivers.zmq_driver import zmq_serializer
|
||||
from oslo_messaging._drivers.zmq_driver import zmq_target
|
||||
|
||||
zmq = zmq_async.import_zmq()
|
||||
|
||||
|
||||
class PublisherBackend(base_proxy.BaseBackendMatcher):
|
||||
|
||||
def __init__(self, conf, context):
|
||||
poller = zmq_async.get_poller(native_zmq=conf.rpc_zmq_native)
|
||||
super(PublisherBackend, self).__init__(conf, poller, context)
|
||||
self.backend = self.context.socket(zmq.PUB)
|
||||
self.backend.bind(zmq_target.get_ipc_address_fanout(conf))
|
||||
|
||||
def redirect_to_backend(self, message):
|
||||
target_pos = zmq_serializer.MESSAGE_CALL_TARGET_POSITION + 1
|
||||
msg = message[target_pos:]
|
||||
self.backend.send_multipart(msg)
|
||||
|
||||
def close(self):
|
||||
self.backend.close()
|
@ -1,72 +0,0 @@
|
||||
# Copyright 2015 Mirantis, Inc.
|
||||
#
|
||||
# Licensed under the Apache License, Version 2.0 (the "License"); you may
|
||||
# not use this file except in compliance with the License. You may obtain
|
||||
# a copy of the License at
|
||||
#
|
||||
# http://www.apache.org/licenses/LICENSE-2.0
|
||||
#
|
||||
# Unless required by applicable law or agreed to in writing, software
|
||||
# distributed under the License is distributed on an "AS IS" BASIS, WITHOUT
|
||||
# WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. See the
|
||||
# License for the specific language governing permissions and limitations
|
||||
# under the License.
|
||||
|
||||
import logging
|
||||
|
||||
import oslo_messaging._drivers.zmq_driver.broker.zmq_base_proxy as base_proxy
|
||||
from oslo_messaging._drivers.zmq_driver.broker import zmq_call_proxy
|
||||
from oslo_messaging._drivers.zmq_driver.broker import zmq_fanout_proxy
|
||||
from oslo_messaging._drivers.zmq_driver import zmq_async
|
||||
from oslo_messaging._drivers.zmq_driver import zmq_serializer
|
||||
from oslo_messaging._i18n import _LI
|
||||
|
||||
LOG = logging.getLogger(__name__)
|
||||
|
||||
|
||||
class UniversalProxy(base_proxy.BaseProxy):
|
||||
|
||||
def __init__(self, conf, context):
|
||||
super(UniversalProxy, self).__init__(conf, context)
|
||||
self.poller = zmq_async.get_poller(
|
||||
native_zmq=conf.rpc_zmq_native)
|
||||
self.tcp_frontend = zmq_call_proxy.FrontendTcpRouter(
|
||||
conf, context, poller=self.poller)
|
||||
self.backend_matcher = BackendMatcher(
|
||||
conf, context, poller=self.poller)
|
||||
call = zmq_serializer.CALL_TYPE
|
||||
self.call_backend = self.backend_matcher.backends[call]
|
||||
LOG.info(_LI("Starting universal-proxy thread"))
|
||||
|
||||
def run(self):
|
||||
message, socket = self.poller.poll(self.conf.rpc_poll_timeout)
|
||||
if message is None:
|
||||
return
|
||||
|
||||
LOG.info(_LI("Received message at universal proxy: %s") % str(message))
|
||||
|
||||
if socket == self.tcp_frontend.frontend:
|
||||
self.backend_matcher.redirect_to_backend(message)
|
||||
else:
|
||||
self.tcp_frontend.redirect_outgoing_reply(message)
|
||||
|
||||
def stop(self):
|
||||
self.poller.close()
|
||||
super(UniversalProxy, self).stop()
|
||||
self.tcp_frontend.close()
|
||||
self.backend_matcher.close()
|
||||
|
||||
|
||||
class BackendMatcher(base_proxy.BaseBackendMatcher):
|
||||
|
||||
def __init__(self, conf, context, poller=None):
|
||||
super(BackendMatcher, self).__init__(conf, poller, context)
|
||||
direct_backend = zmq_call_proxy.DealerBackend(conf, context, poller)
|
||||
self.backends[zmq_serializer.CALL_TYPE] = direct_backend
|
||||
self.backends[zmq_serializer.CAST_TYPE] = direct_backend
|
||||
fanout_backend = zmq_fanout_proxy.PublisherBackend(conf, context)
|
||||
self.backends[zmq_serializer.FANOUT_TYPE] = fanout_backend
|
||||
|
||||
def redirect_to_backend(self, message):
|
||||
message_type = zmq_serializer.get_msg_type(message)
|
||||
self.backends[message_type].redirect_to_backend(message)
|
@ -1,37 +0,0 @@
|
||||
# Copyright 2015 Mirantis, Inc.
|
||||
#
|
||||
# Licensed under the Apache License, Version 2.0 (the "License"); you may
|
||||
# not use this file except in compliance with the License. You may obtain
|
||||
# a copy of the License at
|
||||
#
|
||||
# http://www.apache.org/licenses/LICENSE-2.0
|
||||
#
|
||||
# Unless required by applicable law or agreed to in writing, software
|
||||
# distributed under the License is distributed on an "AS IS" BASIS, WITHOUT
|
||||
# WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. See the
|
||||
# License for the specific language governing permissions and limitations
|
||||
# under the License.
|
||||
|
||||
import abc
|
||||
|
||||
import six
|
||||
|
||||
|
||||
@six.add_metaclass(abc.ABCMeta)
|
||||
class ConsumerBase(object):
|
||||
|
||||
def __init__(self, listener, conf, zmq_poller, context):
|
||||
self.listener = listener
|
||||
self.conf = conf
|
||||
self.poller = zmq_poller
|
||||
self.context = context
|
||||
self.sockets_per_target = {}
|
||||
|
||||
def cleanup(self):
|
||||
if self.sockets_per_target:
|
||||
for socket in self.sockets_per_target.values():
|
||||
socket.close()
|
||||
|
||||
@abc.abstractmethod
|
||||
def listen(self, target):
|
||||
"""Listen for target"""
|
@ -1,74 +0,0 @@
|
||||
# Copyright 2015 Mirantis, Inc.
|
||||
#
|
||||
# Licensed under the Apache License, Version 2.0 (the "License"); you may
|
||||
# not use this file except in compliance with the License. You may obtain
|
||||
# a copy of the License at
|
||||
#
|
||||
# http://www.apache.org/licenses/LICENSE-2.0
|
||||
#
|
||||
# Unless required by applicable law or agreed to in writing, software
|
||||
# distributed under the License is distributed on an "AS IS" BASIS, WITHOUT
|
||||
# WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. See the
|
||||
# License for the specific language governing permissions and limitations
|
||||
# under the License.
|
||||
|
||||
|
||||
import logging
|
||||
|
||||
import six
|
||||
|
||||
from oslo_messaging._drivers import base
|
||||
from oslo_messaging._drivers.zmq_driver.rpc.server import zmq_base_consumer
|
||||
from oslo_messaging._drivers.zmq_driver import zmq_async
|
||||
from oslo_messaging._drivers.zmq_driver import zmq_target as topic_utils
|
||||
from oslo_messaging._i18n import _LE
|
||||
|
||||
|
||||
LOG = logging.getLogger(__name__)
|
||||
|
||||
zmq = zmq_async.import_zmq()
|
||||
|
||||
|
||||
class ZmqFanoutMessage(base.IncomingMessage):
|
||||
|
||||
def __init__(self, listener, context, message, socket, poller):
|
||||
super(ZmqFanoutMessage, self).__init__(listener, context, message)
|
||||
poller.resume_polling(socket)
|
||||
|
||||
def reply(self, reply=None, failure=None, log_failure=True):
|
||||
"""Reply is not needed for fanout(cast) messages"""
|
||||
|
||||
def acknowledge(self):
|
||||
pass
|
||||
|
||||
def requeue(self):
|
||||
pass
|
||||
|
||||
|
||||
class FanoutConsumer(zmq_base_consumer.ConsumerBase):
|
||||
|
||||
def _receive_message(self, socket):
|
||||
try:
|
||||
topic = socket.recv_string()
|
||||
assert topic is not None, 'Bad format: Topic is expected'
|
||||
msg_id = socket.recv_string()
|
||||
assert msg_id is not None, 'Bad format: message ID expected'
|
||||
context = socket.recv_json()
|
||||
message = socket.recv_json()
|
||||
LOG.debug("[Server] REP Received message %s" % str(message))
|
||||
incoming = ZmqFanoutMessage(self.listener, context, message,
|
||||
socket, self.poller)
|
||||
return incoming
|
||||
except zmq.ZMQError as e:
|
||||
LOG.error(_LE("Receiving message failed ... {}"), e)
|
||||
|
||||
def listen(self, target):
|
||||
topic = topic_utils.target_to_str(target)
|
||||
ipc_address = topic_utils.get_ipc_address_fanout(self.conf)
|
||||
sub_socket = self.context.socket(zmq.SUB)
|
||||
sub_socket.connect(ipc_address)
|
||||
if six.PY3:
|
||||
sub_socket.setsockopt_string(zmq.SUBSCRIBE, str(topic))
|
||||
else:
|
||||
sub_socket.setsockopt(zmq.SUBSCRIBE, str(topic))
|
||||
self.poller.register(sub_socket, self._receive_message)
|
Loading…
Reference in New Issue