Merge pull request #190 from pikers/ems_to_bidir_streaming

Ems to bidir streaming
ems_hotfixes
goodboy 2021-06-10 08:45:44 -04:00 committed by GitHub
commit 689bc0cde0
No known key found for this signature in database
GPG Key ID: 4AEE18F83AFDEB23
10 changed files with 1349 additions and 677 deletions

View File

@ -102,7 +102,9 @@ async def open_pikerd(
assert _services is None assert _services is None
# XXX: this may open a root actor as well # XXX: this may open a root actor as well
async with tractor.open_root_actor( async with (
tractor.open_root_actor(
# passed through to ``open_root_actor`` # passed through to ``open_root_actor``
arbiter_addr=_tractor_kwargs['arbiter_addr'], arbiter_addr=_tractor_kwargs['arbiter_addr'],
name=_root_dname, name=_root_dname,
@ -113,10 +115,10 @@ async def open_pikerd(
# TODO: eventually we should be able to avoid # TODO: eventually we should be able to avoid
# having the root have more then permissions to # having the root have more then permissions to
# spawn other specialized daemons I think? # spawn other specialized daemons I think?
# enable_modules=[__name__],
enable_modules=_root_modules, enable_modules=_root_modules,
) as _,
) as _, tractor.open_nursery() as actor_nursery: tractor.open_nursery() as actor_nursery,
):
async with trio.open_nursery() as service_nursery: async with trio.open_nursery() as service_nursery:
# setup service mngr singleton instance # setup service mngr singleton instance
@ -137,6 +139,7 @@ async def open_pikerd(
async def maybe_open_runtime( async def maybe_open_runtime(
loglevel: Optional[str] = None, loglevel: Optional[str] = None,
**kwargs, **kwargs,
) -> None: ) -> None:
""" """
Start the ``tractor`` runtime (a root actor) if none exists. Start the ``tractor`` runtime (a root actor) if none exists.
@ -159,6 +162,7 @@ async def maybe_open_runtime(
async def maybe_open_pikerd( async def maybe_open_pikerd(
loglevel: Optional[str] = None, loglevel: Optional[str] = None,
**kwargs, **kwargs,
) -> Union[tractor._portal.Portal, Services]: ) -> Union[tractor._portal.Portal, Services]:
"""If no ``pikerd`` daemon-root-actor can be found start it and """If no ``pikerd`` daemon-root-actor can be found start it and
yield up (we should probably figure out returning a portal to self yield up (we should probably figure out returning a portal to self
@ -197,6 +201,66 @@ _data_mods = [
] ]
class Brokerd:
locks = defaultdict(trio.Lock)
@asynccontextmanager
async def maybe_spawn_daemon(
service_name: str,
spawn_func: Callable,
spawn_args: dict[str, Any],
loglevel: Optional[str] = None,
**kwargs,
) -> tractor.Portal:
"""
If no ``service_name`` daemon-actor can be found,
spawn one in a local subactor and return a portal to it.
"""
if loglevel:
get_console_log(loglevel)
# serialize access to this section to avoid
# 2 or more tasks racing to create a daemon
lock = Brokerd.locks[service_name]
await lock.acquire()
# attach to existing brokerd if possible
async with tractor.find_actor(service_name) as portal:
if portal is not None:
lock.release()
yield portal
return
# ask root ``pikerd`` daemon to spawn the daemon we need if
# pikerd is not live we now become the root of the
# process tree
async with maybe_open_pikerd(
loglevel=loglevel,
**kwargs,
) as pikerd_portal:
if pikerd_portal is None:
# we are root so spawn brokerd directly in our tree
# the root nursery is accessed through process global state
await spawn_func(**spawn_args)
else:
await pikerd_portal.run(
spawn_func,
**spawn_args,
)
async with tractor.wait_for_actor(service_name) as portal:
lock.release()
yield portal
async def spawn_brokerd( async def spawn_brokerd(
brokername: str, brokername: str,
@ -205,8 +269,6 @@ async def spawn_brokerd(
) -> tractor._portal.Portal: ) -> tractor._portal.Portal:
from .data import _setup_persistent_brokerd
log.info(f'Spawning {brokername} broker daemon') log.info(f'Spawning {brokername} broker daemon')
brokermod = get_brokermod(brokername) brokermod = get_brokermod(brokername)
@ -226,13 +288,9 @@ async def spawn_brokerd(
**tractor_kwargs **tractor_kwargs
) )
# TODO: so i think this is the perfect use case for supporting
# a cross-actor async context manager api instead of this
# shoort-and-forget task spawned in the root nursery, we'd have an
# async exit stack that we'd register the `portal.open_context()`
# call with and then have the ability to unwind the call whenevs.
# non-blocking setup of brokerd service nursery # non-blocking setup of brokerd service nursery
from .data import _setup_persistent_brokerd
await _services.open_remote_ctx( await _services.open_remote_ctx(
portal, portal,
_setup_persistent_brokerd, _setup_persistent_brokerd,
@ -242,10 +300,6 @@ async def spawn_brokerd(
return dname return dname
class Brokerd:
locks = defaultdict(trio.Lock)
@asynccontextmanager @asynccontextmanager
async def maybe_spawn_brokerd( async def maybe_spawn_brokerd(
@ -253,57 +307,24 @@ async def maybe_spawn_brokerd(
loglevel: Optional[str] = None, loglevel: Optional[str] = None,
**kwargs, **kwargs,
) -> tractor._portal.Portal: ) -> tractor.Portal:
""" '''Helper to spawn a brokerd service.
If no ``brokerd.{brokername}`` daemon-actor can be found,
spawn one in a local subactor and return a portal to it.
""" '''
if loglevel: async with maybe_spawn_daemon(
get_console_log(loglevel)
dname = f'brokerd.{brokername}' f'brokerd.{brokername}',
spawn_func=spawn_brokerd,
# serialize access to this section to avoid spawn_args={'brokername': brokername, 'loglevel': loglevel},
# 2 or more tasks racing to create a daemon
lock = Brokerd.locks[brokername]
await lock.acquire()
# attach to existing brokerd if possible
async with tractor.find_actor(dname) as portal:
if portal is not None:
lock.release()
yield portal
return
# ask root ``pikerd`` daemon to spawn the daemon we need if
# pikerd is not live we now become the root of the
# process tree
async with maybe_open_pikerd(
loglevel=loglevel, loglevel=loglevel,
**kwargs, **kwargs,
) as pikerd_portal:
if pikerd_portal is None: ) as portal:
# we are root so spawn brokerd directly in our tree
# the root nursery is accessed through process global state
await spawn_brokerd(brokername, loglevel=loglevel)
else:
await pikerd_portal.run(
spawn_brokerd,
brokername=brokername,
loglevel=loglevel,
)
async with tractor.wait_for_actor(dname) as portal:
lock.release()
yield portal yield portal
async def spawn_emsd( async def spawn_emsd(
brokername: str,
loglevel: Optional[str] = None, loglevel: Optional[str] = None,
**extra_tractor_kwargs **extra_tractor_kwargs
@ -314,10 +335,10 @@ async def spawn_emsd(
""" """
log.info('Spawning emsd') log.info('Spawning emsd')
# TODO: raise exception when _services == None?
global _services global _services
assert _services
await _services.actor_n.start_actor( portal = await _services.actor_n.start_actor(
'emsd', 'emsd',
enable_modules=[ enable_modules=[
'piker.clearing._ems', 'piker.clearing._ems',
@ -327,4 +348,34 @@ async def spawn_emsd(
debug_mode=_services.debug_mode, # set by pikerd flag debug_mode=_services.debug_mode, # set by pikerd flag
**extra_tractor_kwargs **extra_tractor_kwargs
) )
# non-blocking setup of clearing service
from .clearing._ems import _setup_persistent_emsd
await _services.open_remote_ctx(
portal,
_setup_persistent_emsd,
)
return 'emsd' return 'emsd'
@asynccontextmanager
async def maybe_open_emsd(
brokername: str,
loglevel: Optional[str] = None,
**kwargs,
) -> tractor._portal.Portal: # noqa
async with maybe_spawn_daemon(
'emsd',
spawn_func=spawn_emsd,
spawn_args={'loglevel': loglevel},
loglevel=loglevel,
**kwargs,
) as portal:
yield portal

View File

@ -25,7 +25,7 @@ from contextlib import asynccontextmanager
from dataclasses import asdict from dataclasses import asdict
from datetime import datetime from datetime import datetime
from functools import partial from functools import partial
from typing import List, Dict, Any, Tuple, Optional, AsyncIterator, Callable from typing import List, Dict, Any, Tuple, Optional, AsyncIterator
import asyncio import asyncio
from pprint import pformat from pprint import pformat
import inspect import inspect
@ -39,7 +39,8 @@ import tractor
from async_generator import aclosing from async_generator import aclosing
from ib_insync.wrapper import RequestError from ib_insync.wrapper import RequestError
from ib_insync.contract import Contract, ContractDetails, Option from ib_insync.contract import Contract, ContractDetails, Option
from ib_insync.order import Order from ib_insync.order import Order, Trade, OrderStatus
from ib_insync.objects import Fill, Execution
from ib_insync.ticker import Ticker from ib_insync.ticker import Ticker
from ib_insync.objects import Position from ib_insync.objects import Position
import ib_insync as ibis import ib_insync as ibis
@ -53,6 +54,12 @@ from .._daemon import maybe_spawn_brokerd
from ..data._source import from_df from ..data._source import from_df
from ..data._sharedmem import ShmArray from ..data._sharedmem import ShmArray
from ._util import SymbolNotFound, NoData from ._util import SymbolNotFound, NoData
from ..clearing._messages import (
BrokerdOrder, BrokerdOrderAck, BrokerdStatus,
BrokerdPosition, BrokerdCancel,
BrokerdFill,
# BrokerdError,
)
log = get_logger(__name__) log = get_logger(__name__)
@ -472,7 +479,7 @@ class Client:
# XXX: by default 0 tells ``ib_insync`` methods that there is no # XXX: by default 0 tells ``ib_insync`` methods that there is no
# existing order so ask the client to create a new one (which it # existing order so ask the client to create a new one (which it
# seems to do by allocating an int counter - collision prone..) # seems to do by allocating an int counter - collision prone..)
brid: int = None, reqid: int = None,
) -> int: ) -> int:
"""Place an order and return integer request id provided by client. """Place an order and return integer request id provided by client.
@ -488,7 +495,7 @@ class Client:
trade = self.ib.placeOrder( trade = self.ib.placeOrder(
contract, contract,
Order( Order(
orderId=brid or 0, # stupid api devs.. orderId=reqid or 0, # stupid api devs..
action=action.upper(), # BUY/SELL action=action.upper(), # BUY/SELL
orderType='LMT', orderType='LMT',
lmtPrice=price, lmtPrice=price,
@ -582,6 +589,7 @@ class Client:
self, self,
to_trio: trio.abc.SendChannel, to_trio: trio.abc.SendChannel,
) -> None: ) -> None:
# connect error msgs # connect error msgs
def push_err( def push_err(
reqId: int, reqId: int,
@ -589,13 +597,16 @@ class Client:
errorString: str, errorString: str,
contract: Contract, contract: Contract,
) -> None: ) -> None:
log.error(errorString) log.error(errorString)
try: try:
to_trio.send_nowait(( to_trio.send_nowait((
'error', 'error',
# error "object" # error "object"
{'reqid': reqId, {'reqid': reqId,
'message': errorString, 'reason': errorString,
'contract': contract} 'contract': contract}
)) ))
except trio.BrokenResourceError: except trio.BrokenResourceError:
@ -635,6 +646,8 @@ async def _aio_get_client(
"""Return an ``ib_insync.IB`` instance wrapped in our client API. """Return an ``ib_insync.IB`` instance wrapped in our client API.
Client instances are cached for later use. Client instances are cached for later use.
TODO: consider doing this with a ctx mngr eventually?
""" """
# first check cache for existing client # first check cache for existing client
@ -738,7 +751,7 @@ async def _trio_run_client_method(
class _MethodProxy: class _MethodProxy:
def __init__( def __init__(
self, self,
portal: tractor._portal.Portal portal: tractor.Portal
) -> None: ) -> None:
self._portal = portal self._portal = portal
@ -755,7 +768,12 @@ class _MethodProxy:
) )
def get_client_proxy(portal, target=Client) -> _MethodProxy: def get_client_proxy(
portal: tractor.Portal,
target=Client,
) -> _MethodProxy:
proxy = _MethodProxy(portal) proxy = _MethodProxy(portal)
@ -843,7 +861,7 @@ async def get_bars(
end_dt: str = "", end_dt: str = "",
) -> (dict, np.ndarray): ) -> (dict, np.ndarray):
_err = None _err: Optional[Exception] = None
fails = 0 fails = 0
for _ in range(2): for _ in range(2):
@ -880,12 +898,12 @@ async def get_bars(
raise NoData(f'Symbol: {sym}') raise NoData(f'Symbol: {sym}')
break break
else: else:
log.exception( log.exception(
"Data query rate reached: Press `ctrl-alt-f`" "Data query rate reached: Press `ctrl-alt-f`"
"in TWS" "in TWS"
) )
print(_err)
# TODO: should probably create some alert on screen # TODO: should probably create some alert on screen
# and then somehow get that to trigger an event here # and then somehow get that to trigger an event here
@ -932,7 +950,7 @@ async def backfill_bars(
if fails is None or fails > 1: if fails is None or fails > 1:
break break
if out is (None, None): if out == (None, None):
# could be trying to retreive bars over weekend # could be trying to retreive bars over weekend
# TODO: add logic here to handle tradable hours and only grab # TODO: add logic here to handle tradable hours and only grab
# valid bars in the range # valid bars in the range
@ -1183,112 +1201,213 @@ def pack_position(pos: Position) -> Dict[str, Any]:
else: else:
symbol = con.symbol symbol = con.symbol
return { return BrokerdPosition(
'broker': 'ib', broker='ib',
'account': pos.account, account=pos.account,
'symbol': symbol, symbol=symbol,
'currency': con.currency, currency=con.currency,
'size': float(pos.position), size=float(pos.position),
'avg_price': float(pos.avgCost) / float(con.multiplier or 1.0), avg_price=float(pos.avgCost) / float(con.multiplier or 1.0),
}
@tractor.msg.pub(
send_on_connect={'local_trades': 'start'}
) )
async def stream_trades(
async def handle_order_requests(
ems_order_stream: tractor.MsgStream,
) -> None:
# request_msg: dict
async for request_msg in ems_order_stream:
log.info(f'Received order request {request_msg}')
action = request_msg['action']
if action in {'buy', 'sell'}:
# validate
order = BrokerdOrder(**request_msg)
# call our client api to submit the order
reqid = await _trio_run_client_method(
method='submit_limit',
oid=order.oid,
symbol=order.symbol,
price=order.price,
action=order.action,
size=order.size,
# XXX: by default 0 tells ``ib_insync`` methods that
# there is no existing order so ask the client to create
# a new one (which it seems to do by allocating an int
# counter - collision prone..)
reqid=order.reqid,
)
# deliver ack that order has been submitted to broker routing
await ems_order_stream.send(
BrokerdOrderAck(
# ems order request id
oid=order.oid,
# broker specific request id
reqid=reqid,
time_ns=time.time_ns(),
).dict()
)
elif action == 'cancel':
msg = BrokerdCancel(**request_msg)
await _trio_run_client_method(
method='submit_cancel',
reqid=msg.reqid
)
else:
log.error(f'Unknown order command: {request_msg}')
@tractor.context
async def trades_dialogue(
ctx: tractor.Context,
loglevel: str = None, loglevel: str = None,
get_topics: Callable = None,
) -> AsyncIterator[Dict[str, Any]]: ) -> AsyncIterator[Dict[str, Any]]:
# XXX: required to propagate ``tractor`` loglevel to piker logging # XXX: required to propagate ``tractor`` loglevel to piker logging
get_console_log(loglevel or tractor.current_actor().loglevel) get_console_log(loglevel or tractor.current_actor().loglevel)
stream = await _trio_run_client_method( ib_trade_events_stream = await _trio_run_client_method(
method='recv_trade_updates', method='recv_trade_updates',
) )
# deliver positions to subscriber before anything else # deliver positions to subscriber before anything else
positions = await _trio_run_client_method(method='positions') positions = await _trio_run_client_method(method='positions')
all_positions = {}
for pos in positions: for pos in positions:
yield {'local_trades': ('position', pack_position(pos))} msg = pack_position(pos)
all_positions[msg.symbol] = msg.dict()
await ctx.started(all_positions)
action_map = {'BOT': 'buy', 'SLD': 'sell'} action_map = {'BOT': 'buy', 'SLD': 'sell'}
async for event_name, item in stream: async with (
ctx.open_stream() as ems_stream,
trio.open_nursery() as n,
):
# start order request handler **before** local trades event loop
n.start_soon(handle_order_requests, ems_stream)
async for event_name, item in ib_trade_events_stream:
# XXX: begin normalization of nonsense ib_insync internal # XXX: begin normalization of nonsense ib_insync internal
# object-state tracking representations... # object-state tracking representations...
if event_name == 'status': if event_name == 'status':
# unwrap needed data from ib_insync internal objects # unwrap needed data from ib_insync internal types
trade = item trade: Trade = item
status = trade.orderStatus status: OrderStatus = trade.orderStatus
# skip duplicate filled updates - we get the deats # skip duplicate filled updates - we get the deats
# from the execution details event # from the execution details event
msg = { msg = BrokerdStatus(
'reqid': trade.order.orderId,
'status': status.status,
'filled': status.filled,
'reason': status.whyHeld,
reqid=trade.order.orderId,
time_ns=time.time_ns(), # cuz why not
status=status.status.lower(), # force lower case
filled=status.filled,
reason=status.whyHeld,
# this seems to not be necessarily up to date in the # this seems to not be necessarily up to date in the
# execDetails event.. so we have to send it here I guess? # execDetails event.. so we have to send it here I guess?
'remaining': status.remaining, remaining=status.remaining,
}
broker_details={'name': 'ib'},
)
elif event_name == 'fill': elif event_name == 'fill':
# for wtv reason this is a separate event type
# from IB, not sure why it's needed other then for extra
# complexity and over-engineering :eyeroll:.
# we may just end up dropping these events (or
# translating them to ``Status`` msgs) if we can
# show the equivalent status events are no more latent.
# unpack ib_insync types
# pep-0526 style:
# https://www.python.org/dev/peps/pep-0526/#global-and-local-variable-annotations
trade: Trade
fill: Fill
trade, fill = item trade, fill = item
execu = fill.execution execu: Execution = fill.execution
msg = { # TODO: normalize out commissions details?
'reqid': execu.orderId, details = {
'execid': execu.execId, 'contract': asdict(fill.contract),
'execution': asdict(fill.execution),
# supposedly IB server fill time 'commissions': asdict(fill.commissionReport),
'broker_time': execu.time, # converted to float by us 'broker_time': execu.time, # supposedly IB server fill time
# ns from main TCP handler by us inside ``ib_insync`` override 'name': 'ib',
'time': fill.time,
'time_ns': time.time_ns(), # cuz why not
'action': action_map[execu.side],
'size': execu.shares,
'price': execu.price,
} }
msg = BrokerdFill(
# should match the value returned from `.submit_limit()`
reqid=execu.orderId,
time_ns=time.time_ns(), # cuz why not
action=action_map[execu.side],
size=execu.shares,
price=execu.price,
broker_details=details,
# XXX: required by order mode currently
broker_time=details['broker_time'],
)
elif event_name == 'error': elif event_name == 'error':
msg = item
err: dict = item
# f$#$% gawd dammit insync.. # f$#$% gawd dammit insync..
con = msg['contract'] con = err['contract']
if isinstance(con, Contract): if isinstance(con, Contract):
msg['contract'] = asdict(con) err['contract'] = asdict(con)
if msg['reqid'] == -1: if err['reqid'] == -1:
log.error(pformat(msg)) log.error(f'TWS external order error:\n{pformat(err)}')
# don't forward, it's pointless.. # don't forward for now, it's unecessary.. but if we wanted to,
# msg = BrokerdError(**err)
continue continue
elif event_name == 'position': elif event_name == 'position':
msg = pack_position(item) msg = pack_position(item)
if msg.get('reqid', 0) < -1: if getattr(msg, 'reqid', 0) < -1:
# it's a trade event generated by TWS usage. # it's a trade event generated by TWS usage.
log.warning(f"TWS triggered trade:\n{pformat(msg)}") log.warning(f"TWS triggered trade:\n{pformat(msg)}")
msg['reqid'] = 'tws-' + str(-1 * msg['reqid']) msg.reqid = 'tws-' + str(-1 * msg.reqid)
# mark msg as from "external system" # mark msg as from "external system"
# TODO: probably something better then this.. # TODO: probably something better then this.. and start
# considering multiplayer/group trades tracking
msg['external'] = True msg['external'] = True
yield {'remote_trades': (event_name, msg)}
continue continue
yield {'local_trades': (event_name, msg)} # XXX: we always serialize to a dict for msgpack
# translations, ideally we can move to an msgspec (or other)
# encoder # that can be enabled in ``tractor`` ahead of
# time so we can pass through the message types directly.
await ems_stream.send(msg.dict())
@tractor.context @tractor.context

View File

@ -19,32 +19,23 @@ Orders and execution client API.
""" """
from contextlib import asynccontextmanager from contextlib import asynccontextmanager
from typing import Dict, Tuple, List from typing import Dict
from pprint import pformat from pprint import pformat
from dataclasses import dataclass, field from dataclasses import dataclass, field
import trio import trio
import tractor import tractor
# import msgspec
from ..data._source import Symbol from ..data._source import Symbol
from ..log import get_logger from ..log import get_logger
from ._ems import _emsd_main from ._ems import _emsd_main
from .._daemon import maybe_open_emsd
from ._messages import Order, Cancel
log = get_logger(__name__) log = get_logger(__name__)
# class Order(msgspec.Struct):
# action: str
# price: float
# size: float
# symbol: str
# brokers: List[str]
# oid: str
# exec_mode: str
@dataclass @dataclass
class OrderBook: class OrderBook:
"""Buy-side (client-side ?) order book ctl and tracking. """Buy-side (client-side ?) order book ctl and tracking.
@ -62,31 +53,34 @@ class OrderBook:
_to_ems: trio.abc.SendChannel _to_ems: trio.abc.SendChannel
_from_order_book: trio.abc.ReceiveChannel _from_order_book: trio.abc.ReceiveChannel
_sent_orders: Dict[str, dict] = field(default_factory=dict) _sent_orders: Dict[str, Order] = field(default_factory=dict)
_ready_to_receive: trio.Event = trio.Event() _ready_to_receive: trio.Event = trio.Event()
def send( def send(
self, self,
uuid: str, uuid: str,
symbol: str, symbol: str,
brokers: List[str], brokers: list[str],
price: float, price: float,
size: float, size: float,
action: str, action: str,
exec_mode: str, exec_mode: str,
) -> dict: ) -> dict:
cmd = { msg = Order(
'action': action, action=action,
'price': price, price=price,
'size': size, size=size,
'symbol': symbol, symbol=symbol,
'brokers': brokers, brokers=brokers,
'oid': uuid, oid=uuid,
'exec_mode': exec_mode, # dark or live exec_mode=exec_mode, # dark or live
} )
self._sent_orders[uuid] = cmd
self._to_ems.send_nowait(cmd) self._sent_orders[uuid] = msg
return cmd self._to_ems.send_nowait(msg.dict())
return msg
def update( def update(
self, self,
@ -94,29 +88,29 @@ class OrderBook:
**data: dict, **data: dict,
) -> dict: ) -> dict:
cmd = self._sent_orders[uuid] cmd = self._sent_orders[uuid]
cmd.update(data) msg = cmd.dict()
self._sent_orders[uuid] = cmd msg.update(data)
self._to_ems.send_nowait(cmd) self._sent_orders[uuid] = Order(**msg)
self._to_ems.send_nowait(msg)
return cmd return cmd
def cancel(self, uuid: str) -> bool: def cancel(self, uuid: str) -> bool:
"""Cancel an order (or alert) from the EMS. """Cancel an order (or alert) in the EMS.
""" """
cmd = self._sent_orders[uuid] cmd = self._sent_orders[uuid]
msg = { msg = Cancel(
'action': 'cancel', oid=uuid,
'oid': uuid, symbol=cmd.symbol,
'symbol': cmd['symbol'], )
} self._to_ems.send_nowait(msg.dict())
self._to_ems.send_nowait(msg)
_orders: OrderBook = None _orders: OrderBook = None
def get_orders( def get_orders(
emsd_uid: Tuple[str, str] = None emsd_uid: tuple[str, str] = None
) -> OrderBook: ) -> OrderBook:
"""" """"
OrderBook singleton factory per actor. OrderBook singleton factory per actor.
@ -136,7 +130,14 @@ def get_orders(
return _orders return _orders
async def send_order_cmds(symbol_key: str): # TODO: we can get rid of this relay loop once we move
# order_mode inputs to async code!
async def relay_order_cmds_from_sync_code(
symbol_key: str,
to_ems_stream: tractor.MsgStream,
) -> None:
""" """
Order streaming task: deliver orders transmitted from UI Order streaming task: deliver orders transmitted from UI
to downstream consumers. to downstream consumers.
@ -156,16 +157,15 @@ async def send_order_cmds(symbol_key: str):
book = get_orders() book = get_orders()
orders_stream = book._from_order_book orders_stream = book._from_order_book
# signal that ems connection is up and ready
book._ready_to_receive.set()
async for cmd in orders_stream: async for cmd in orders_stream:
print(cmd) print(cmd)
if cmd['symbol'] == symbol_key: if cmd['symbol'] == symbol_key:
# send msg over IPC / wire # send msg over IPC / wire
log.info(f'Send order cmd:\n{pformat(cmd)}') log.info(f'Send order cmd:\n{pformat(cmd)}')
yield cmd await to_ems_stream.send(cmd)
else: else:
# XXX BRUTAL HACKZORZES !!! # XXX BRUTAL HACKZORZES !!!
# re-insert for another consumer # re-insert for another consumer
@ -174,36 +174,12 @@ async def send_order_cmds(symbol_key: str):
book._to_ems.send_nowait(cmd) book._to_ems.send_nowait(cmd)
@asynccontextmanager
async def maybe_open_emsd(
brokername: str,
) -> tractor._portal.Portal: # noqa
async with tractor.find_actor('emsd') as portal:
if portal is not None:
yield portal
return
# ask remote daemon tree to spawn it
from .._daemon import spawn_emsd
async with tractor.find_actor('pikerd') as portal:
assert portal
name = await portal.run(
spawn_emsd,
brokername=brokername,
)
async with tractor.wait_for_actor(name) as portal:
yield portal
@asynccontextmanager @asynccontextmanager
async def open_ems( async def open_ems(
broker: str, broker: str,
symbol: Symbol, symbol: Symbol,
) -> None:
) -> (OrderBook, tractor.MsgStream, dict):
"""Spawn an EMS daemon and begin sending orders and receiving """Spawn an EMS daemon and begin sending orders and receiving
alerts. alerts.
@ -237,32 +213,31 @@ async def open_ems(
- 'broker_filled' - 'broker_filled'
""" """
actor = tractor.current_actor()
# wait for service to connect back to us signalling # wait for service to connect back to us signalling
# ready for order commands # ready for order commands
book = get_orders() book = get_orders()
async with maybe_open_emsd(broker) as portal: async with maybe_open_emsd(broker) as portal:
async with portal.open_stream_from( async with (
# connect to emsd
portal.open_context(
_emsd_main, _emsd_main,
client_actor_name=actor.name,
broker=broker, broker=broker,
symbol=symbol.key, symbol=symbol.key,
) as trades_stream: ) as (ctx, positions),
with trio.fail_after(10):
await book._ready_to_receive.wait()
try: # open 2-way trade command stream
yield book, trades_stream ctx.open_stream() as trades_stream,
):
async with trio.open_nursery() as n:
n.start_soon(
relay_order_cmds_from_sync_code,
symbol.key,
trades_stream
)
finally: yield book, trades_stream, positions
# TODO: we want to eventually keep this up (by having
# the exec loop keep running in the pikerd tree) but for
# now we have to kill the context to avoid backpressure
# build-up on the shm write loop.
with trio.CancelScope(shield=True):
await trades_stream.aclose()

View File

@ -21,19 +21,22 @@ In da suit parlances: "Execution management systems"
from pprint import pformat from pprint import pformat
import time import time
from dataclasses import dataclass, field from dataclasses import dataclass, field
from typing import ( from typing import AsyncIterator, Callable, Any
AsyncIterator, Dict, Callable, Tuple,
)
from bidict import bidict from bidict import bidict
from pydantic import BaseModel
import trio import trio
from trio_typing import TaskStatus
import tractor import tractor
from .. import data from .. import data
from ..log import get_logger from ..log import get_logger
from ..data._normalize import iterticks from ..data._normalize import iterticks
from ._paper_engine import PaperBoi, simulate_fills from . import _paper_engine as paper
from ._messages import (
Status, Order,
BrokerdCancel, BrokerdOrder, BrokerdOrderAck, BrokerdStatus,
BrokerdFill, BrokerdError, BrokerdPosition,
)
log = get_logger(__name__) log = get_logger(__name__)
@ -89,11 +92,11 @@ class _DarkBook:
broker: str broker: str
# levels which have an executable action (eg. alert, order, signal) # levels which have an executable action (eg. alert, order, signal)
orders: Dict[ orders: dict[
str, # symbol str, # symbol
Dict[ dict[
str, # uuid str, # uuid
Tuple[ tuple[
Callable[[float], bool], # predicate Callable[[float], bool], # predicate
str, # name str, # name
dict, # cmd / msg type dict, # cmd / msg type
@ -102,22 +105,14 @@ class _DarkBook:
] = field(default_factory=dict) ] = field(default_factory=dict)
# tracks most recent values per symbol each from data feed # tracks most recent values per symbol each from data feed
lasts: Dict[ lasts: dict[
Tuple[str, str], tuple[str, str],
float float
] = field(default_factory=dict) ] = field(default_factory=dict)
# mapping of broker order ids to piker ems ids # mapping of piker ems order ids to current brokerd order flow message
_broker2ems_ids: Dict[str, str] = field(default_factory=bidict) _ems_entries: dict[str, str] = field(default_factory=dict)
_ems2brokerd_ids: dict[str, str] = field(default_factory=bidict)
_books: Dict[str, _DarkBook] = {}
def get_dark_book(broker: str) -> _DarkBook:
global _books
return _books.setdefault(broker, _DarkBook(broker))
# XXX: this is in place to prevent accidental positions that are too # XXX: this is in place to prevent accidental positions that are too
@ -127,13 +122,17 @@ def get_dark_book(broker: str) -> _DarkBook:
_DEFAULT_SIZE: float = 1.0 _DEFAULT_SIZE: float = 1.0
async def execute_triggers( async def clear_dark_triggers(
brokerd_orders_stream: tractor.MsgStream,
ems_client_order_stream: tractor.MsgStream,
quote_stream: tractor.ReceiveMsgStream, # noqa
broker: str, broker: str,
symbol: str, symbol: str,
stream: 'tractor.ReceiveStream', # noqa
ctx: tractor.Context,
client: 'Client', # noqa
book: _DarkBook, book: _DarkBook,
) -> None: ) -> None:
"""Core dark order trigger loop. """Core dark order trigger loop.
@ -143,7 +142,7 @@ async def execute_triggers(
""" """
# this stream may eventually contain multiple symbols # this stream may eventually contain multiple symbols
# XXX: optimize this for speed! # XXX: optimize this for speed!
async for quotes in stream: async for quotes in quote_stream:
# TODO: numba all this! # TODO: numba all this!
@ -179,10 +178,15 @@ async def execute_triggers(
# majority of iterations will be non-matches # majority of iterations will be non-matches
continue continue
action = cmd['action'] action: str = cmd['action']
symbol: str = cmd['symbol']
if action != 'alert': if action == 'alert':
# executable order submission # nothing to do but relay a status
# message back to the requesting ems client
resp = 'alert_triggered'
else: # executable order submission
# submit_price = price + price*percent_away # submit_price = price + price*percent_away
submit_price = price + abs_diff_away submit_price = price + abs_diff_away
@ -191,47 +195,56 @@ async def execute_triggers(
f'Dark order triggered for price {price}\n' f'Dark order triggered for price {price}\n'
f'Submitting order @ price {submit_price}') f'Submitting order @ price {submit_price}')
reqid = await client.submit_limit( msg = BrokerdOrder(
action=cmd['action'],
oid=oid, oid=oid,
time_ns=time.time_ns(),
# this is a brand new order request for the # this **creates** new order request for the
# underlying broker so we set out "broker request # underlying broker so we set a "broker
# id" (brid) as nothing so that the broker # request id" (``reqid`` kwarg) to ``None``
# client knows that we aren't trying to modify # so that the broker client knows that we
# an existing order. # aren't trying to modify an existing
brid=None, # order-request and instead create a new one.
reqid=None,
symbol=sym, symbol=sym,
action=cmd['action'],
price=submit_price, price=submit_price,
size=cmd['size'], size=cmd['size'],
) )
await brokerd_orders_stream.send(msg.dict())
# register broker request id to ems id # mark this entry as having sent an order
book._broker2ems_ids[reqid] = oid # request. the entry will be replaced once the
# target broker replies back with
# a ``BrokerdOrderAck`` msg including the
# allocated unique ``BrokerdOrderAck.reqid`` key
# generated by the broker's own systems.
book._ems_entries[oid] = msg
else: # our internal status value for client-side
# alerts have no broker request id # triggered "dark orders"
reqid = '' resp = 'dark_triggered'
resp = { msg = Status(
'resp': 'dark_executed', oid=oid, # ems order id
'time_ns': time.time_ns(), resp=resp,
'trigger_price': price, time_ns=time.time_ns(),
'cmd': cmd, # original request message symbol=symbol,
trigger_price=price,
'broker_reqid': reqid, broker_details={'name': broker},
'broker': broker,
'oid': oid, # piker order id
} cmd=cmd, # original request message
).dict()
# remove exec-condition from set # remove exec-condition from set
log.info(f'removing pred for {oid}') log.info(f'removing pred for {oid}')
execs.pop(oid) execs.pop(oid)
await ctx.send_yield(resp) await ems_client_order_stream.send(msg)
else: # condition scan loop complete else: # condition scan loop complete
log.debug(f'execs are {execs}') log.debug(f'execs are {execs}')
@ -241,78 +254,6 @@ async def execute_triggers(
# print(f'execs scan took: {time.time() - start}') # print(f'execs scan took: {time.time() - start}')
async def exec_loop(
ctx: tractor.Context,
feed: 'Feed', # noqa
broker: str,
symbol: str,
_exec_mode: str,
task_status: TaskStatus[dict] = trio.TASK_STATUS_IGNORED,
) -> AsyncIterator[dict]:
"""Main scan loop for order execution conditions and submission
to brokers.
"""
# XXX: this should be initial price quote from target provider
first_quote = await feed.receive()
book = get_dark_book(broker)
book.lasts[(broker, symbol)] = first_quote[symbol]['last']
# TODO: wrap this in a more re-usable general api
client_factory = getattr(feed.mod, 'get_client_proxy', None)
if client_factory is not None and _exec_mode != 'paper':
# we have an order API for this broker
client = client_factory(feed._brokerd_portal)
else:
# force paper mode
log.warning(f'Entering paper trading mode for {broker}')
client = PaperBoi(
broker,
*trio.open_memory_channel(100),
_buys={},
_sells={},
_reqids={},
)
# for paper mode we need to mock this trades response feed
# so we pass a duck-typed feed-looking mem chan which is fed
# fill and submission events from the exec loop
feed._trade_stream = client.trade_stream
# init the trades stream
client._to_trade_stream.send_nowait({'local_trades': 'start'})
_exec_mode = 'paper'
# return control to parent task
task_status.started((first_quote, feed, client))
stream = feed.stream
async with trio.open_nursery() as n:
n.start_soon(
execute_triggers,
broker,
symbol,
stream,
ctx,
client,
book
)
if _exec_mode == 'paper':
# TODO: make this an actual broadcast channels as in:
# https://github.com/python-trio/trio/issues/987
n.start_soon(simulate_fills, stream, client)
# TODO: lots of cases still to handle # TODO: lots of cases still to handle
# XXX: right now this is very very ad-hoc to IB # XXX: right now this is very very ad-hoc to IB
# - short-sale but securities haven't been located, in this case we # - short-sale but securities haven't been located, in this case we
@ -323,11 +264,13 @@ async def exec_loop(
# reqId 1550: Order held while securities are located.'), # reqId 1550: Order held while securities are located.'),
# status='PreSubmitted', message='')], # status='PreSubmitted', message='')],
async def process_broker_trades( async def translate_and_relay_brokerd_events(
ctx: tractor.Context,
feed: 'Feed', # noqa broker: str,
ems_client_order_stream: tractor.MsgStream,
brokerd_trades_stream: tractor.MsgStream,
book: _DarkBook, book: _DarkBook,
task_status: TaskStatus[dict] = trio.TASK_STATUS_IGNORED,
) -> AsyncIterator[dict]: ) -> AsyncIterator[dict]:
"""Trades update loop - receive updates from broker, convert """Trades update loop - receive updates from broker, convert
to EMS responses, transmit to ordering client(s). to EMS responses, transmit to ordering client(s).
@ -347,59 +290,93 @@ async def process_broker_trades(
{'presubmitted', 'submitted', 'cancelled', 'inactive'} {'presubmitted', 'submitted', 'cancelled', 'inactive'}
""" """
broker = feed.mod.name async for brokerd_msg in brokerd_trades_stream:
# TODO: make this a context name = brokerd_msg['name']
# in the paper engine case this is just a mem receive channel
async with feed.receive_trades_data() as trades_stream:
first = await trades_stream.__anext__() log.info(f'Received broker trade event:\n{pformat(brokerd_msg)}')
# startup msg expected as first from broker backend
assert first['local_trades'] == 'start'
task_status.started()
async for event in trades_stream:
name, msg = event['local_trades']
log.info(f'Received broker trade event:\n{pformat(msg)}')
if name == 'position': if name == 'position':
msg['resp'] = 'position'
# relay through # relay through position msgs immediately
await ctx.send_yield(msg) await ems_client_order_stream.send(
BrokerdPosition(**brokerd_msg).dict()
)
continue continue
# Get the broker (order) request id, this **must** be normalized # Get the broker (order) request id, this **must** be normalized
# into messaging provided by the broker backend # into messaging provided by the broker backend
reqid = msg['reqid'] reqid = brokerd_msg['reqid']
# make response packet to EMS client(s) # all piker originated requests will have an ems generated oid field
oid = book._broker2ems_ids.get(reqid) oid = brokerd_msg.get(
'oid',
book._ems2brokerd_ids.inverse.get(reqid)
)
if oid is None: if oid is None:
# XXX: paper clearing special cases
# paper engine race case: ``Client.submit_limit()`` hasn't # paper engine race case: ``Client.submit_limit()`` hasn't
# returned yet and provided an output reqid to register # returned yet and provided an output reqid to register
# locally, so we need to retreive the oid that was already # locally, so we need to retreive the oid that was already
# packed at submission since we already know it ahead of # packed at submission since we already know it ahead of
# time # time
paper = msg.get('paper_info') paper = brokerd_msg['broker_details'].get('paper_info')
if paper: if paper:
# paperboi keeps the ems id up front
oid = paper['oid'] oid = paper['oid']
else: else:
msg.get('external') # may be an order msg specified as "external" to the
if not msg: # piker ems flow (i.e. generated by some other
log.error(f"Unknown trade event {event}") # external broker backend client (like tws for ib)
ext = brokerd_msg.get('external')
if ext:
log.error(f"External trade event {ext}")
continue
else:
# check for existing live flow entry
entry = book._ems_entries.get(oid)
# initial response to brokerd order request
if name == 'ack':
# register the brokerd request id (that was generated
# / created internally by the broker backend) with our
# local ems order id for reverse lookup later.
# a ``BrokerdOrderAck`` **must** be sent after an order
# request in order to establish this id mapping.
book._ems2brokerd_ids[oid] = reqid
# new order which has not yet be registered into the
# local ems book, insert it now and handle 2 cases:
# - the order has previously been requested to be
# cancelled by the ems controlling client before we
# received this ack, in which case we relay that cancel
# signal **asap** to the backend broker
if entry.action == 'cancel':
# assign newly providerd broker backend request id
entry.reqid = reqid
# tell broker to cancel immediately
await brokerd_trades_stream.send(entry.dict())
# - the order is now active and will be mirrored in
# our book -> registered as live flow
else:
# update the flow with the ack msg
book._ems_entries[oid] = BrokerdOrderAck(**brokerd_msg)
continue continue
resp = { # a live flow now exists
'resp': None, # placeholder oid = entry.oid
'oid': oid
} resp = None
broker_details = {}
if name in ( if name in (
'error', 'error',
@ -413,10 +390,10 @@ async def process_broker_trades(
# about. In most default situations, with composed orders # about. In most default situations, with composed orders
# (ex. brackets), most brokers seem to use a oca policy. # (ex. brackets), most brokers seem to use a oca policy.
message = msg['message'] msg = BrokerdError(**brokerd_msg)
# XXX should we make one when it's blank? # XXX should we make one when it's blank?
log.error(pformat(message)) log.error(pformat(msg))
# TODO: getting this bs, prolly need to handle status messages # TODO: getting this bs, prolly need to handle status messages
# 'Market data farm connection is OK:usfarm.nj' # 'Market data farm connection is OK:usfarm.nj'
@ -444,15 +421,15 @@ async def process_broker_trades(
# - Inactive (reject or cancelled but not by trader) # - Inactive (reject or cancelled but not by trader)
# everyone doin camel case # everyone doin camel case
status = msg['status'].lower() msg = BrokerdStatus(**brokerd_msg)
if status == 'filled': if msg.status == 'filled':
# conditional execution is fully complete, no more # conditional execution is fully complete, no more
# fills for the noted order # fills for the noted order
if not msg['remaining']: if not msg.remaining:
resp['resp'] = 'broker_executed' resp = 'broker_executed'
log.info(f'Execution for {oid} is complete!') log.info(f'Execution for {oid} is complete!')
@ -461,81 +438,134 @@ async def process_broker_trades(
log.info(f'{broker} filled {msg}') log.info(f'{broker} filled {msg}')
else: else:
# one of (submitted, cancelled) # one of {submitted, cancelled}
resp['resp'] = 'broker_' + status resp = 'broker_' + msg.status
# pass the BrokerdStatus msg inside the broker details field
broker_details = msg.dict()
elif name in ( elif name in (
'fill', 'fill',
): ):
msg = BrokerdFill(**brokerd_msg)
# proxy through the "fill" result(s) # proxy through the "fill" result(s)
resp['resp'] = 'broker_filled' resp = 'broker_filled'
resp.update(msg) broker_details = msg.dict()
log.info(f'\nFill for {oid} cleared with:\n{pformat(resp)}') log.info(f'\nFill for {oid} cleared with:\n{pformat(resp)}')
# respond to requesting client else:
await ctx.send_yield(resp) raise ValueError(f'Brokerd message {brokerd_msg} is invalid')
# Create and relay response status message
# to requesting EMS client
await ems_client_order_stream.send(
Status(
oid=oid,
resp=resp,
time_ns=time.time_ns(),
broker_reqid=reqid,
brokerd_msg=broker_details,
).dict()
)
async def process_order_cmds( async def process_client_order_cmds(
ctx: tractor.Context,
cmd_stream: 'tractor.ReceiveStream', # noqa client_order_stream: tractor.MsgStream, # noqa
brokerd_order_stream: tractor.MsgStream,
symbol: str, symbol: str,
feed: 'Feed', # noqa feed: 'Feed', # noqa
client: 'Client', # noqa
dark_book: _DarkBook, dark_book: _DarkBook,
) -> None: ) -> None:
async for cmd in cmd_stream: # cmd: dict
async for cmd in client_order_stream:
log.info(f'Received order cmd:\n{pformat(cmd)}') log.info(f'Received order cmd:\n{pformat(cmd)}')
action = cmd['action'] action = cmd['action']
oid = cmd['oid'] oid = cmd['oid']
reqid = dark_book._ems2brokerd_ids.inverse.get(oid)
brid = dark_book._broker2ems_ids.inverse.get(oid) live_entry = dark_book._ems_entries.get(oid)
# TODO: can't wait for this stuff to land in 3.10 # TODO: can't wait for this stuff to land in 3.10
# https://www.python.org/dev/peps/pep-0636/#going-to-the-cloud-mappings # https://www.python.org/dev/peps/pep-0636/#going-to-the-cloud-mappings
if action in ('cancel',): if action in ('cancel',):
# check for live-broker order # check for live-broker order
if brid: if live_entry:
msg = BrokerdCancel(
oid=oid,
reqid=reqid or live_entry.reqid,
time_ns=time.time_ns(),
)
# send cancel to brokerd immediately!
log.info("Submitting cancel for live order") log.info("Submitting cancel for live order")
await client.submit_cancel(reqid=brid)
# NOTE: cancel response will be relayed back in messages
# from corresponding broker
await brokerd_order_stream.send(msg.dict())
else:
# this might be a cancel for an order that hasn't been
# acked yet by a brokerd, so register a cancel for when
# the order ack does show up later
dark_book._ems_entries[oid] = msg
# check for EMS active exec # check for EMS active exec
else:
try: try:
# remove from dark book clearing
dark_book.orders[symbol].pop(oid, None) dark_book.orders[symbol].pop(oid, None)
await ctx.send_yield({ # tell client side that we've cancelled the
'resp': 'dark_cancelled', # dark-trigger order
'oid': oid await client_order_stream.send(
}) Status(
resp='dark_cancelled',
oid=oid,
time_ns=time.time_ns(),
).dict()
)
except KeyError: except KeyError:
log.exception(f'No dark order for {symbol}?') log.exception(f'No dark order for {symbol}?')
# TODO: 3.10 struct-pattern matching and unpacking here
elif action in ('alert', 'buy', 'sell',): elif action in ('alert', 'buy', 'sell',):
sym = cmd['symbol'] msg = Order(**cmd)
trigger_price = cmd['price']
size = cmd['size']
brokers = cmd['brokers']
exec_mode = cmd['exec_mode']
broker = brokers[0] sym = msg.symbol
trigger_price = msg.price
size = msg.size
exec_mode = msg.exec_mode
broker = msg.brokers[0]
if exec_mode == 'live' and action in ('buy', 'sell',): if exec_mode == 'live' and action in ('buy', 'sell',):
# register broker id for ems id if live_entry is not None:
order_id = await client.submit_limit(
# sanity check on emsd id
assert live_entry.oid == oid
# if we already had a broker order id then
# this is likely an order update commmand.
log.info(
f"Modifying live {broker} order: {live_entry.reqid}")
msg = BrokerdOrder(
oid=oid, # no ib support for oids... oid=oid, # no ib support for oids...
time_ns=time.time_ns(),
# if this is None, creates a new order # if this is None, creates a new order
# otherwise will modify any existing one # otherwise will modify any existing one
brid=brid, reqid=reqid,
symbol=sym, symbol=sym,
action=action, action=action,
@ -543,25 +573,24 @@ async def process_order_cmds(
size=size, size=size,
) )
if brid: # send request to backend
assert dark_book._broker2ems_ids[brid] == oid
# if we already had a broker order id then
# this is likely an order update commmand.
log.info(f"Modifying order: {brid}")
else:
dark_book._broker2ems_ids[order_id] = oid
# XXX: the trades data broker response loop # XXX: the trades data broker response loop
# (``process_broker_trades()`` above) will # (``translate_and_relay_brokerd_events()`` above) will
# handle sending the ems side acks back to # handle relaying the ems side responses back to
# the cmd sender from here # the client/cmd sender from this request
log.info(f'Sending live order to {broker}:\n{pformat(msg)}')
await brokerd_order_stream.send(msg.dict())
# an immediate response should be brokerd ack with order
# id but we register our request as part of the flow
dark_book._ems_entries[oid] = msg
elif exec_mode in ('dark', 'paper') or ( elif exec_mode in ('dark', 'paper') or (
action in ('alert') action in ('alert')
): ):
# submit order to local EMS # submit order to local EMS book and scan loop,
# effectively a local clearing engine, which
# scans for conditions and triggers matching executions
# Auto-gen scanner predicate: # Auto-gen scanner predicate:
# we automatically figure out what the alert check # we automatically figure out what the alert check
@ -595,8 +624,10 @@ async def process_order_cmds(
abs_diff_away = 0 abs_diff_away = 0
# submit execution/order to EMS scan loop # submit execution/order to EMS scan loop
# FYI: this may result in an override of an existing
# NOTE: this may result in an override of an existing
# dark book entry if the order id already exists # dark book entry if the order id already exists
dark_book.orders.setdefault( dark_book.orders.setdefault(
sym, {} sym, {}
)[oid] = ( )[oid] = (
@ -606,25 +637,32 @@ async def process_order_cmds(
percent_away, percent_away,
abs_diff_away abs_diff_away
) )
# TODO: if the predicate resolves immediately send the
# execution to the broker asap? Or no?
# ack-response that order is live in EMS if action == 'alert':
await ctx.send_yield({ resp = 'alert_submitted'
'resp': 'dark_submitted', else:
'oid': oid resp = 'dark_submitted'
})
await client_order_stream.send(
Status(
resp=resp,
oid=oid,
time_ns=time.time_ns(),
).dict()
)
@tractor.stream @tractor.context
async def _emsd_main( async def _emsd_main(
ctx: tractor.Context, ctx: tractor.Context,
client_actor_name: str,
broker: str, broker: str,
symbol: str, symbol: str,
_mode: str = 'dark', # ('paper', 'dark', 'live') _exec_mode: str = 'dark', # ('paper', 'dark', 'live')
loglevel: str = 'info',
) -> None: ) -> None:
"""EMS (sub)actor entrypoint providing the '''EMS (sub)actor entrypoint providing the
execution management (micro)service which conducts broker execution management (micro)service which conducts broker
order control on behalf of clients. order control on behalf of clients.
@ -638,66 +676,183 @@ async def _emsd_main(
received in a stream from that client actor and then responses are received in a stream from that client actor and then responses are
streamed back up to the original calling task in the same client. streamed back up to the original calling task in the same client.
The task tree is: The primary ``emsd`` task tree is:
- ``_emsd_main()``: - ``_emsd_main()``:
accepts order cmds, registers execs with exec loop sets up brokerd feed, order feed with ems client, trades dialogue with
brokderd trading api.
|
- ``clear_dark_triggers()``:
run (dark order) conditions on inputs and trigger brokerd "live"
order submissions.
|
- ``translate_and_relay_brokerd_events()``:
accept normalized trades responses from brokerd, process and
relay to ems client(s); this is a effectively a "trade event
reponse" proxy-broker.
|
- ``process_client_order_cmds()``:
accepts order cmds from requesting piker clients, registers
execs with exec loop
- ``exec_loop()``: '''
run (dark) conditions on inputs and trigger broker submissions global _router
dark_book = _router.get_dark_book(broker)
- ``process_broker_trades()``: # TODO: would be nice if in tractor we can require either a ctx arg,
accept normalized trades responses, process and relay to ems client(s) # or a named arg with ctx in it and a type annotation of
# tractor.Context instead of strictly requiring a ctx arg.
ems_ctx = ctx
""" cached_feed = _router.feeds.get((broker, symbol))
from ._client import send_order_cmds if cached_feed:
# TODO: use cached feeds per calling-actor
dark_book = get_dark_book(broker) log.warning(f'Opening duplicate feed for {(broker, symbol)}')
# spawn one task per broker feed # spawn one task per broker feed
async with trio.open_nursery() as n: async with (
trio.open_nursery() as n,
# TODO: eventually support N-brokers # TODO: eventually support N-brokers
async with data.open_feed( data.open_feed(
broker, broker,
[symbol], [symbol],
loglevel='info', loglevel=loglevel,
) as feed: ) as feed,
):
if not cached_feed:
_router.feeds[(broker, symbol)] = feed
# get a portal back to the client # XXX: this should be initial price quote from target provider
async with tractor.wait_for_actor(client_actor_name) as portal: first_quote = await feed.receive()
# connect back to the calling actor (the one that is # open a stream with the brokerd backend for order
# acting as an EMS client and will submit orders) to # flow dialogue
# receive requests pushed over a tractor stream
# using (for now) an async generator. book = _router.get_dark_book(broker)
async with portal.open_stream_from( book.lasts[(broker, symbol)] = first_quote[symbol]['last']
send_order_cmds,
symbol_key=symbol, trades_endpoint = getattr(feed.mod, 'trades_dialogue', None)
) as order_stream: portal = feed._brokerd_portal
if trades_endpoint is None or _exec_mode == 'paper':
# for paper mode we need to mock this trades response feed
# so we load bidir stream to a new sub-actor running a
# paper-simulator clearing engine.
# load the paper trading engine
_exec_mode = 'paper'
log.warning(f'Entering paper trading mode for {broker}')
# load the paper trading engine as a subactor of this emsd
# actor to simulate the real IPC load it'll have when also
# pulling data from feeds
open_trades_endpoint = paper.open_paperboi(
broker=broker,
symbol=symbol,
loglevel=loglevel,
)
else:
# open live brokerd trades endpoint
open_trades_endpoint = portal.open_context(
trades_endpoint,
loglevel=loglevel,
)
async with (
open_trades_endpoint as (brokerd_ctx, positions),
brokerd_ctx.open_stream() as brokerd_trades_stream,
):
# signal to client that we're started
# TODO: we could eventually send back **all** brokerd
# positions here?
await ems_ctx.started(positions)
# establish 2-way stream with requesting order-client and
# begin handling inbound order requests and updates
async with ems_ctx.open_stream() as ems_client_order_stream:
# trigger scan and exec loop
n.start_soon(
clear_dark_triggers,
brokerd_trades_stream,
ems_client_order_stream,
feed.stream,
# start the condition scan loop
quote, feed, client = await n.start(
exec_loop,
ctx,
feed,
broker, broker,
symbol, symbol,
_mode, book
) )
await n.start( # begin processing order events from the target brokerd backend
process_broker_trades, # by receiving order submission response messages,
ctx, # normalizing them to EMS messages and relaying back to
feed, # the piker order client.
n.start_soon(
translate_and_relay_brokerd_events,
broker,
ems_client_order_stream,
brokerd_trades_stream,
dark_book, dark_book,
) )
# start inbound order request processing # start inbound (from attached client) order request processing
await process_order_cmds( await process_client_order_cmds(
ctx, ems_client_order_stream,
order_stream, brokerd_trades_stream,
symbol, symbol,
feed, feed,
client,
dark_book, dark_book,
) )
class _Router(BaseModel):
'''Order router which manages per-broker dark books, alerts,
and clearing related data feed management.
'''
nursery: trio.Nursery
feeds: dict[tuple[str, str], Any] = {}
books: dict[str, _DarkBook] = {}
class Config:
arbitrary_types_allowed = True
underscore_attrs_are_private = False
def get_dark_book(
self,
brokername: str,
) -> _DarkBook:
return self.books.setdefault(brokername, _DarkBook(brokername))
_router: _Router = None
@tractor.context
async def _setup_persistent_emsd(
ctx: tractor.Context,
) -> None:
global _router
# open a root "service nursery" for the ``emsd`` actor
async with trio.open_nursery() as service_nursery:
_router = _Router(nursery=service_nursery)
# TODO: send back the full set of persistent orders/execs persistent
await ctx.started()
# we pin this task to keep the feeds manager active until the
# parent actor decides to tear it down
await trio.sleep_forever()

View File

@ -0,0 +1,238 @@
# piker: trading gear for hackers
# Copyright (C) Tyler Goodlet (in stewardship for piker0)
# This program is free software: you can redistribute it and/or modify
# it under the terms of the GNU Affero General Public License as published by
# the Free Software Foundation, either version 3 of the License, or
# (at your option) any later version.
# This program is distributed in the hope that it will be useful,
# but WITHOUT ANY WARRANTY; without even the implied warranty of
# MERCHANTABILITY or FITNESS FOR A PARTICULAR PURPOSE. See the
# GNU Affero General Public License for more details.
# You should have received a copy of the GNU Affero General Public License
# along with this program. If not, see <https://www.gnu.org/licenses/>.
"""
Clearing system messagingn types and protocols.
"""
from typing import Optional, Union
# TODO: try out just encoding/send direction for now?
# import msgspec
from pydantic import BaseModel
# Client -> emsd
class Cancel(BaseModel):
'''Cancel msg for removing a dark (ems triggered) or
broker-submitted (live) trigger/order.
'''
action: str = 'cancel'
oid: str # uuid4
symbol: str
class Order(BaseModel):
action: str # {'buy', 'sell', 'alert'}
# internal ``emdsd`` unique "order id"
oid: str # uuid4
symbol: str
price: float
size: float
brokers: list[str]
# Assigned once initial ack is received
# ack_time_ns: Optional[int] = None
# determines whether the create execution
# will be submitted to the ems or directly to
# the backend broker
exec_mode: str # {'dark', 'live', 'paper'}
# Client <- emsd
# update msgs from ems which relay state change info
# from the active clearing engine.
class Status(BaseModel):
name: str = 'status'
oid: str # uuid4
time_ns: int
# {
# 'dark_submitted',
# 'dark_cancelled',
# 'dark_triggered',
# 'broker_submitted',
# 'broker_cancelled',
# 'broker_executed',
# 'broker_filled',
# 'alert_submitted',
# 'alert_triggered',
# 'position',
# }
resp: str # "response", see above
# symbol: str
# trigger info
trigger_price: Optional[float] = None
# price: float
# broker: Optional[str] = None
# this maps normally to the ``BrokerdOrder.reqid`` below, an id
# normally allocated internally by the backend broker routing system
broker_reqid: Optional[Union[int, str]] = None
# for relaying backend msg data "through" the ems layer
brokerd_msg: dict = {}
# emsd -> brokerd
# requests *sent* from ems to respective backend broker daemon
class BrokerdCancel(BaseModel):
action: str = 'cancel'
oid: str # piker emsd order id
time_ns: int
# "broker request id": broker specific/internal order id if this is
# None, creates a new order otherwise if the id is valid the backend
# api must modify the existing matching order. If the broker allows
# for setting a unique order id then this value will be relayed back
# on the emsd order request stream as the ``BrokerdOrderAck.reqid``
# field
reqid: Optional[Union[int, str]] = None
class BrokerdOrder(BaseModel):
action: str # {buy, sell}
oid: str
time_ns: int
# "broker request id": broker specific/internal order id if this is
# None, creates a new order otherwise if the id is valid the backend
# api must modify the existing matching order. If the broker allows
# for setting a unique order id then this value will be relayed back
# on the emsd order request stream as the ``BrokerdOrderAck.reqid``
# field
reqid: Optional[Union[int, str]] = None
symbol: str # symbol.<providername> ?
price: float
size: float
# emsd <- brokerd
# requests *received* to ems from broker backend
class BrokerdOrderAck(BaseModel):
'''Immediate reponse to a brokerd order request providing
the broker specifci unique order id.
'''
name: str = 'ack'
# defined and provided by backend
reqid: Union[int, str]
# emsd id originally sent in matching request msg
oid: str
class BrokerdStatus(BaseModel):
name: str = 'status'
reqid: Union[int, str]
time_ns: int
# {
# 'submitted',
# 'cancelled',
# 'executed',
# }
status: str
filled: float = 0.0
reason: str = ''
remaining: float = 0.0
# XXX: better design/name here?
# flag that can be set to indicate a message for an order
# event that wasn't originated by piker's emsd (eg. some external
# trading system which does it's own order control but that you
# might want to "track" using piker UIs/systems).
external: bool = False
# XXX: not required schema as of yet
broker_details: dict = {
'name': '',
}
class BrokerdFill(BaseModel):
'''A single message indicating a "fill-details" event from the broker
if avaiable.
'''
name: str = 'fill'
reqid: Union[int, str]
time_ns: int
# order exeuction related
action: str
size: float
price: float
broker_details: dict = {} # meta-data (eg. commisions etc.)
# brokerd timestamp required for order mode arrow placement on x-axis
# TODO: maybe int if we force ns?
# we need to normalize this somehow since backends will use their
# own format and likely across many disparate epoch clocks...
broker_time: float
class BrokerdError(BaseModel):
'''Optional error type that can be relayed to emsd for error handling.
This is still a TODO thing since we're not sure how to employ it yet.
'''
name: str = 'error'
reqid: Union[int, str]
symbol: str
reason: str
broker_details: dict = {}
class BrokerdPosition(BaseModel):
'''Position update event from brokerd.
'''
name: str = 'position'
broker: str
account: str
symbol: str
currency: str
size: float
avg_price: float

View File

@ -18,17 +18,28 @@
Fake trading for forward testing. Fake trading for forward testing.
""" """
from contextlib import asynccontextmanager
from datetime import datetime from datetime import datetime
from operator import itemgetter from operator import itemgetter
import time import time
from typing import Tuple, Optional from typing import Tuple, Optional, Callable
import uuid import uuid
from bidict import bidict from bidict import bidict
import trio import trio
import tractor
from dataclasses import dataclass from dataclasses import dataclass
from .. import data
from ..data._normalize import iterticks from ..data._normalize import iterticks
from ..log import get_logger
from ._messages import (
BrokerdCancel, BrokerdOrder, BrokerdOrderAck, BrokerdStatus,
BrokerdFill,
)
log = get_logger(__name__)
@dataclass @dataclass
@ -41,8 +52,8 @@ class PaperBoi:
""" """
broker: str broker: str
_to_trade_stream: trio.abc.SendChannel
trade_stream: trio.abc.ReceiveChannel ems_trades_stream: tractor.MsgStream
# map of paper "live" orders which be used # map of paper "live" orders which be used
# to simulate fills based on paper engine settings # to simulate fills based on paper engine settings
@ -61,20 +72,20 @@ class PaperBoi:
price: float, price: float,
action: str, action: str,
size: float, size: float,
brid: Optional[str], reqid: Optional[str],
) -> int: ) -> int:
"""Place an order and return integer request id provided by client. """Place an order and return integer request id provided by client.
""" """
is_modify: bool = False
if brid is None: if reqid is None:
reqid = str(uuid.uuid4()) reqid = str(uuid.uuid4())
else: else:
# order is already existing, this is a modify # order is already existing, this is a modify
(oid, symbol, action, old_price) = self._reqids[brid] (oid, symbol, action, old_price) = self._reqids[reqid]
assert old_price != price assert old_price != price
reqid = brid is_modify = True
# register order internally # register order internally
self._reqids[reqid] = (oid, symbol, action, price) self._reqids[reqid] = (oid, symbol, action, price)
@ -90,22 +101,16 @@ class PaperBoi:
# in the broker trades event processing loop # in the broker trades event processing loop
await trio.sleep(0.05) await trio.sleep(0.05)
await self._to_trade_stream.send({ msg = BrokerdStatus(
status='submitted',
'local_trades': ('status', { reqid=reqid,
broker=self.broker,
'time_ns': time.time_ns(), time_ns=time.time_ns(),
'reqid': reqid, filled=0.0,
reason='paper_trigger',
'status': 'submitted', remaining=size,
'broker': self.broker, )
# 'cmd': cmd, # original request message await self.ems_trades_stream.send(msg.dict())
'paper_info': {
'oid': oid,
},
}),
})
# if we're already a clearing price simulate an immediate fill # if we're already a clearing price simulate an immediate fill
if ( if (
@ -129,7 +134,7 @@ class PaperBoi:
# and trigger by the simulated clearing task normally # and trigger by the simulated clearing task normally
# running ``simulate_fills()``. # running ``simulate_fills()``.
if brid is not None: if is_modify:
# remove any existing order for the old price # remove any existing order for the old price
orders[symbol].pop((oid, old_price)) orders[symbol].pop((oid, old_price))
@ -144,7 +149,6 @@ class PaperBoi:
) -> None: ) -> None:
# TODO: fake market simulation effects # TODO: fake market simulation effects
# await self._to_trade_stream.send(
oid, symbol, action, price = self._reqids[reqid] oid, symbol, action, price = self._reqids[reqid]
if action == 'buy': if action == 'buy':
@ -155,21 +159,14 @@ class PaperBoi:
# TODO: net latency model # TODO: net latency model
await trio.sleep(0.05) await trio.sleep(0.05)
await self._to_trade_stream.send({ msg = BrokerdStatus(
status='cancelled',
'local_trades': ('status', { oid=oid,
reqid=reqid,
'time_ns': time.time_ns(), broker=self.broker,
'oid': oid, time_ns=time.time_ns(),
'reqid': reqid, )
await self.ems_trades_stream.send(msg.dict())
'status': 'cancelled',
'broker': self.broker,
# 'cmd': cmd, # original request message
'paper': True,
}),
})
async def fake_fill( async def fake_fill(
self, self,
@ -191,56 +188,49 @@ class PaperBoi:
# TODO: net latency model # TODO: net latency model
await trio.sleep(0.05) await trio.sleep(0.05)
# the trades stream expects events in the form msg = BrokerdFill(
# {'local_trades': (event_name, msg)}
await self._to_trade_stream.send({
'local_trades': ('fill', { reqid=reqid,
time_ns=time.time_ns(),
'status': 'filled', action=action,
'broker': self.broker, size=size,
# converted to float by us in ib backend price=price,
'broker_time': datetime.now().timestamp(),
'action': action,
'size': size,
'price': price,
'remaining': 0 if order_complete else remaining,
# normally filled by real `brokerd` daemon
'time': time.time_ns(),
'time_ns': time.time_ns(), # cuz why not
# fake ids
'reqid': reqid,
broker_time=datetime.now().timestamp(),
broker_details={
'paper_info': { 'paper_info': {
'oid': oid, 'oid': oid,
}, },
# mocking ib
'name': self.broker + '_paper',
},
)
await self.ems_trades_stream.send(msg.dict())
# XXX: fields we might not need to emulate?
# execution id from broker
# 'execid': execu.execId,
# 'cmd': cmd, # original request message?
}),
})
if order_complete: if order_complete:
await self._to_trade_stream.send({
'local_trades': ('status', { msg = BrokerdStatus(
'reqid': reqid,
'status': 'filled',
'broker': self.broker,
'filled': size,
'remaining': 0 if order_complete else remaining,
# converted to float by us in ib backend reqid=reqid,
'broker_time': datetime.now().timestamp(), time_ns=time.time_ns(),
status='filled',
filled=size,
remaining=0 if order_complete else remaining,
action=action,
size=size,
price=price,
broker_details={
'paper_info': { 'paper_info': {
'oid': oid, 'oid': oid,
}, },
}), 'name': self.broker,
}) },
)
await self.ems_trades_stream.send(msg.dict())
async def simulate_fills( async def simulate_fills(
@ -327,3 +317,145 @@ async def simulate_fills(
else: else:
# prices are iterated in sorted order so we're done # prices are iterated in sorted order so we're done
break break
async def handle_order_requests(
client: PaperBoi,
ems_order_stream: tractor.MsgStream,
) -> None:
# order_request: dict
async for request_msg in ems_order_stream:
action = request_msg['action']
if action in {'buy', 'sell'}:
# validate
order = BrokerdOrder(**request_msg)
# call our client api to submit the order
reqid = await client.submit_limit(
oid=order.oid,
symbol=order.symbol,
price=order.price,
action=order.action,
size=order.size,
# XXX: by default 0 tells ``ib_insync`` methods that
# there is no existing order so ask the client to create
# a new one (which it seems to do by allocating an int
# counter - collision prone..)
reqid=order.reqid,
)
# deliver ack that order has been submitted to broker routing
await ems_order_stream.send(
BrokerdOrderAck(
# ems order request id
oid=order.oid,
# broker specific request id
reqid=reqid,
).dict()
)
elif action == 'cancel':
msg = BrokerdCancel(**request_msg)
await client.submit_cancel(
reqid=msg.reqid
)
else:
log.error(f'Unknown order command: {request_msg}')
@tractor.context
async def trades_dialogue(
ctx: tractor.Context,
broker: str,
symbol: str,
loglevel: str = None,
) -> None:
async with (
data.open_feed(
broker,
[symbol],
loglevel=loglevel,
) as feed,
):
# TODO: load paper positions per broker from .toml config file
# and pass as symbol to position data mapping: ``dict[str, dict]``
# await ctx.started(all_positions)
await ctx.started({})
async with (
ctx.open_stream() as ems_stream,
trio.open_nursery() as n,
):
client = PaperBoi(
broker,
ems_stream,
_buys={},
_sells={},
_reqids={},
)
n.start_soon(handle_order_requests, client, ems_stream)
# paper engine simulator clearing task
await simulate_fills(feed.stream, client)
@asynccontextmanager
async def open_paperboi(
broker: str,
symbol: str,
loglevel: str,
) -> Callable:
'''Spawn a paper engine actor and yield through access to
its context.
'''
service_name = f'paperboi.{broker}'
async with (
tractor.find_actor(service_name) as portal,
tractor.open_nursery() as tn,
):
# only spawn if no paperboi already is up
# (we likely don't need more then one proc for basic
# simulated order clearing)
if portal is None:
portal = await tn.start_actor(
service_name,
enable_modules=[__name__]
)
async with portal.open_context(
trades_dialogue,
broker=broker,
symbol=symbol,
loglevel=loglevel,
) as (ctx, first):
try:
yield ctx, first
finally:
# be sure to tear down the paper service on exit
with trio.CancelScope(shield=True):
await portal.cancel_actor()

View File

@ -9,6 +9,7 @@ import tractor
from ..log import get_console_log, get_logger, colorize_json from ..log import get_console_log, get_logger, colorize_json
from ..brokers import get_brokermod, config from ..brokers import get_brokermod, config
from .._daemon import _tractor_kwargs
log = get_logger('cli') log = get_logger('cli')
@ -101,8 +102,9 @@ def cli(ctx, brokers, loglevel, tl, configdir):
def services(config, tl, names): def services(config, tl, names):
async def list_services(): async def list_services():
async with tractor.get_arbiter( async with tractor.get_arbiter(
*tractor.current_actor()._arb_addr *_tractor_kwargs['arbiter_addr']
) as portal: ) as portal:
registry = await portal.run('self', 'get_registry') registry = await portal.run('self', 'get_registry')
json_d = {} json_d = {}
@ -118,6 +120,7 @@ def services(config, tl, names):
list_services, list_services,
name='service_query', name='service_query',
loglevel=config['loglevel'] if tl else None, loglevel=config['loglevel'] if tl else None,
arbiter_addr=_tractor_kwargs['arbiter_addr'],
) )

View File

@ -64,13 +64,13 @@ class NoBsWs:
async def _connect( async def _connect(
self, self,
tries: int = 10000, tries: int = 1000,
) -> None: ) -> None:
while True: while True:
try: try:
await self._stack.aclose() await self._stack.aclose()
except (DisconnectionTimeout, RuntimeError): except (DisconnectionTimeout, RuntimeError):
await trio.sleep(1) await trio.sleep(0.5)
else: else:
break break
@ -95,7 +95,7 @@ class NoBsWs:
f'{self} connection bail with ' f'{self} connection bail with '
f'{type(err)}...retry attempt {i}' f'{type(err)}...retry attempt {i}'
) )
await trio.sleep(1) await trio.sleep(0.5)
continue continue
else: else:
log.exception('ws connection fail...') log.exception('ws connection fail...')

View File

@ -217,8 +217,8 @@ async def allocate_persistent_feed(
times = shm.array['time'] times = shm.array['time']
delay_s = times[-1] - times[times != times[-1]][-1] delay_s = times[-1] - times[times != times[-1]][-1]
# pass OHLC sample rate in seconds # pass OHLC sample rate in seconds (be sure to use python int type)
init_msg[symbol]['sample_rate'] = delay_s init_msg[symbol]['sample_rate'] = int(delay_s)
# yield back control to starting nursery # yield back control to starting nursery
task_status.started((init_msg, first_quote)) task_status.started((init_msg, first_quote))

View File

@ -127,10 +127,6 @@ class OrderMode:
""" """
line = self.lines.commit_line(uuid) line = self.lines.commit_line(uuid)
req_msg = self.book._sent_orders.get(uuid)
if req_msg:
req_msg['ack_time_ns'] = time.time_ns()
return line return line
def on_fill( def on_fill(
@ -196,8 +192,10 @@ class OrderMode:
def submit_exec( def submit_exec(
self, self,
size: Optional[float] = None, size: Optional[float] = None,
) -> LevelLine: ) -> LevelLine:
"""Send execution order to EMS. """Send execution order to EMS return a level line to
represent the order on a chart.
""" """
# register the "staged" line under the cursor # register the "staged" line under the cursor
@ -226,6 +224,9 @@ class OrderMode:
exec_mode=self._exec_mode, exec_mode=self._exec_mode,
) )
# TODO: update the line once an ack event comes back
# from the EMS!
# make line graphic if order push was # make line graphic if order push was
# sucessful # sucessful
line = self.lines.create_order_line( line = self.lines.create_order_line(
@ -266,14 +267,6 @@ class OrderMode:
price=line.value(), price=line.value(),
) )
# def on_key_press(
# self,
# key:
# mods:
# text: str,
# ) -> None:
# pass
@asynccontextmanager @asynccontextmanager
async def open_order_mode( async def open_order_mode(
@ -320,10 +313,14 @@ async def start_order_mode(
# spawn EMS actor-service # spawn EMS actor-service
async with ( async with (
open_ems(brokername, symbol) as (book, trades_stream), open_ems(brokername, symbol) as (book, trades_stream, positions),
open_order_mode(symbol, chart, book) as order_mode open_order_mode(symbol, chart, book) as order_mode
): ):
# update any exising positions
for sym, msg in positions.items():
order_mode.on_position_update(msg)
def get_index(time: float): def get_index(time: float):
# XXX: not sure why the time is so off here # XXX: not sure why the time is so off here
@ -346,16 +343,15 @@ async def start_order_mode(
fmsg = pformat(msg) fmsg = pformat(msg)
log.info(f'Received order msg:\n{fmsg}') log.info(f'Received order msg:\n{fmsg}')
resp = msg['resp'] name = msg['name']
if name in (
if resp in (
'position', 'position',
): ):
# show line label once order is live # show line label once order is live
order_mode.on_position_update(msg) order_mode.on_position_update(msg)
continue continue
# delete the line from view resp = msg['resp']
oid = msg['oid'] oid = msg['oid']
# response to 'action' request (buy/sell) # response to 'action' request (buy/sell)
@ -378,15 +374,15 @@ async def start_order_mode(
order_mode.on_cancel(oid) order_mode.on_cancel(oid)
elif resp in ( elif resp in (
'dark_executed' 'dark_triggered'
): ):
log.info(f'Dark order triggered for {fmsg}') log.info(f'Dark order triggered for {fmsg}')
# for alerts add a triangle and remove the elif resp in (
# level line 'alert_triggered'
if msg['cmd']['action'] == 'alert': ):
# should only be one "fill" for an alert # should only be one "fill" for an alert
# add a triangle and remove the level line
order_mode.on_fill( order_mode.on_fill(
oid, oid,
price=msg['trigger_price'], price=msg['trigger_price'],
@ -403,12 +399,15 @@ async def start_order_mode(
# each clearing tick is responded individually # each clearing tick is responded individually
elif resp in ('broker_filled',): elif resp in ('broker_filled',):
action = msg['action'] action = book._sent_orders[oid].action
details = msg['brokerd_msg']
# TODO: some kinda progress system # TODO: some kinda progress system
order_mode.on_fill( order_mode.on_fill(
oid, oid,
price=msg['price'], price=details['price'],
arrow_index=get_index(msg['broker_time']),
pointing='up' if action == 'buy' else 'down', pointing='up' if action == 'buy' else 'down',
# TODO: put the actual exchange timestamp
arrow_index=get_index(details['broker_time']),
) )