canvas.py 45 KB

1234567891011121314151617181920212223242526272829303132333435363738394041424344454647484950515253545556575859606162636465666768697071727374757677787980818283848586878889909192939495969798991001011021031041051061071081091101111121131141151161171181191201211221231241251261271281291301311321331341351361371381391401411421431441451461471481491501511521531541551561571581591601611621631641651661671681691701711721731741751761771781791801811821831841851861871881891901911921931941951961971981992002012022032042052062072082092102112122132142152162172182192202212222232242252262272282292302312322332342352362372382392402412422432442452462472482492502512522532542552562572582592602612622632642652662672682692702712722732742752762772782792802812822832842852862872882892902912922932942952962972982993003013023033043053063073083093103113123133143153163173183193203213223233243253263273283293303313323333343353363373383393403413423433443453463473483493503513523533543553563573583593603613623633643653663673683693703713723733743753763773783793803813823833843853863873883893903913923933943953963973983994004014024034044054064074084094104114124134144154164174184194204214224234244254264274284294304314324334344354364374384394404414424434444454464474484494504514524534544554564574584594604614624634644654664674684694704714724734744754764774784794804814824834844854864874884894904914924934944954964974984995005015025035045055065075085095105115125135145155165175185195205215225235245255265275285295305315325335345355365375385395405415425435445455465475485495505515525535545555565575585595605615625635645655665675685695705715725735745755765775785795805815825835845855865875885895905915925935945955965975985996006016026036046056066076086096106116126136146156166176186196206216226236246256266276286296306316326336346356366376386396406416426436446456466476486496506516526536546556566576586596606616626636646656666676686696706716726736746756766776786796806816826836846856866876886896906916926936946956966976986997007017027037047057067077087097107117127137147157167177187197207217227237247257267277287297307317327337347357367377387397407417427437447457467477487497507517527537547557567577587597607617627637647657667677687697707717727737747757767777787797807817827837847857867877887897907917927937947957967977987998008018028038048058068078088098108118128138148158168178188198208218228238248258268278288298308318328338348358368378388398408418428438448458468478488498508518528538548558568578588598608618628638648658668678688698708718728738748758768778788798808818828838848858868878888898908918928938948958968978988999009019029039049059069079089099109119129139149159169179189199209219229239249259269279289299309319329339349359369379389399409419429439449459469479489499509519529539549559569579589599609619629639649659669679689699709719729739749759769779789799809819829839849859869879889899909919929939949959969979989991000100110021003100410051006100710081009101010111012101310141015101610171018101910201021102210231024102510261027102810291030103110321033103410351036103710381039104010411042104310441045104610471048104910501051105210531054105510561057105810591060106110621063106410651066106710681069107010711072107310741075107610771078107910801081108210831084108510861087108810891090109110921093109410951096109710981099110011011102110311041105110611071108110911101111111211131114111511161117111811191120112111221123112411251126112711281129113011311132113311341135113611371138113911401141114211431144114511461147114811491150115111521153115411551156115711581159116011611162116311641165116611671168116911701171117211731174117511761177117811791180118111821183118411851186118711881189119011911192119311941195119611971198119912001201120212031204120512061207120812091210121112121213121412151216121712181219122012211222122312241225122612271228122912301231123212331234123512361237123812391240124112421243124412451246124712481249125012511252125312541255125612571258125912601261126212631264126512661267126812691270127112721273127412751276127712781279128012811282128312841285128612871288128912901291129212931294129512961297129812991300130113021303130413051306130713081309131013111312131313141315131613171318131913201321132213231324132513261327132813291330133113321333133413351336
  1. # -*- coding: utf-8 -*-
  2. """Composing task work-flows.
  3. .. seealso:
  4. You should import these from :mod:`celery` and not this module.
  5. """
  6. from __future__ import absolute_import, unicode_literals
  7. import sys
  8. from collections import MutableSequence, deque
  9. from copy import deepcopy
  10. from functools import partial as _partial, reduce
  11. from operator import itemgetter
  12. from itertools import chain as _chain
  13. from kombu.utils.functional import fxrange, reprcall
  14. from kombu.utils.objects import cached_property
  15. from kombu.utils.uuid import uuid
  16. from vine import barrier
  17. from celery._state import current_app
  18. from celery.five import python_2_unicode_compatible
  19. from celery.local import try_import
  20. from celery.result import GroupResult
  21. from celery.utils import abstract
  22. from celery.utils.functional import (
  23. maybe_list, is_list, _regen, regen, chunks as _chunks,
  24. )
  25. from celery.utils.text import truncate
  26. __all__ = [
  27. 'Signature', 'chain', 'xmap', 'xstarmap', 'chunks',
  28. 'group', 'chord', 'signature', 'maybe_signature',
  29. ]
  30. PY3 = sys.version_info[0] == 3
  31. # json in Python 2.7 borks if dict contains byte keys.
  32. JSON_NEEDS_UNICODE_KEYS = PY3 and not try_import('simplejson')
  33. class _getitem_property(object):
  34. """Attribute -> dict key descriptor.
  35. The target object must support ``__getitem__``,
  36. and optionally ``__setitem__``.
  37. Example:
  38. >>> from collections import defaultdict
  39. >>> class Me(dict):
  40. ... deep = defaultdict(dict)
  41. ...
  42. ... foo = _getitem_property('foo')
  43. ... deep_thing = _getitem_property('deep.thing')
  44. >>> me = Me()
  45. >>> me.foo
  46. None
  47. >>> me.foo = 10
  48. >>> me.foo
  49. 10
  50. >>> me['foo']
  51. 10
  52. >>> me.deep_thing = 42
  53. >>> me.deep_thing
  54. 42
  55. >>> me.deep
  56. defaultdict(<type 'dict'>, {'thing': 42})
  57. """
  58. def __init__(self, keypath, doc=None):
  59. path, _, self.key = keypath.rpartition('.')
  60. self.path = path.split('.') if path else None
  61. self.__doc__ = doc
  62. def _path(self, obj):
  63. return (reduce(lambda d, k: d[k], [obj] + self.path) if self.path
  64. else obj)
  65. def __get__(self, obj, type=None):
  66. if obj is None:
  67. return type
  68. return self._path(obj).get(self.key)
  69. def __set__(self, obj, value):
  70. self._path(obj)[self.key] = value
  71. def maybe_unroll_group(g):
  72. """Unroll group with only one member."""
  73. # Issue #1656
  74. try:
  75. size = len(g.tasks)
  76. except TypeError:
  77. try:
  78. size = g.tasks.__length_hint__()
  79. except (AttributeError, TypeError):
  80. return g
  81. else:
  82. return list(g.tasks)[0] if size == 1 else g
  83. else:
  84. return g.tasks[0] if size == 1 else g
  85. def task_name_from(task):
  86. return getattr(task, 'name', task)
  87. def _upgrade(fields, sig):
  88. """Used by custom signatures in .from_dict, to keep common fields."""
  89. sig.update(chord_size=fields.get('chord_size'))
  90. return sig
  91. @abstract.CallableSignature.register
  92. @python_2_unicode_compatible
  93. class Signature(dict):
  94. """Task Signature.
  95. Class that wraps the arguments and execution options
  96. for a single task invocation.
  97. Used as the parts in a :class:`group` and other constructs,
  98. or to pass tasks around as callbacks while being compatible
  99. with serializers with a strict type subset.
  100. Signatures can also be created from tasks:
  101. - Using the ``.signature()`` method that has the same signature
  102. as ``Task.apply_async``:
  103. .. code-block:: pycon
  104. >>> add.signature(args=(1,), kwargs={'kw': 2}, options={})
  105. - or the ``.s()`` shortcut that works for star arguments:
  106. .. code-block:: pycon
  107. >>> add.s(1, kw=2)
  108. - the ``.s()`` shortcut does not allow you to specify execution options
  109. but there's a chaning `.set` method that returns the signature:
  110. .. code-block:: pycon
  111. >>> add.s(2, 2).set(countdown=10).set(expires=30).delay()
  112. Note:
  113. You should use :func:`~celery.signature` to create new signatures.
  114. The ``Signature`` class is the type returned by that function and
  115. should be used for ``isinstance`` checks for signatures.
  116. See Also:
  117. :ref:`guide-canvas` for the complete guide.
  118. Arguments:
  119. task (Task, str): Either a task class/instance, or the name of a task.
  120. args (Tuple): Positional arguments to apply.
  121. kwargs (Dict): Keyword arguments to apply.
  122. options (Dict): Additional options to :meth:`Task.apply_async`.
  123. Note:
  124. If the first argument is a :class:`dict`, the other
  125. arguments will be ignored and the values in the dict will be used
  126. instead::
  127. >>> s = signature('tasks.add', args=(2, 2))
  128. >>> signature(s)
  129. {'task': 'tasks.add', args=(2, 2), kwargs={}, options={}}
  130. """
  131. TYPES = {}
  132. _app = _type = None
  133. @classmethod
  134. def register_type(cls, subclass, name=None):
  135. cls.TYPES[name or subclass.__name__] = subclass
  136. return subclass
  137. @classmethod
  138. def from_dict(cls, d, app=None):
  139. typ = d.get('subtask_type')
  140. if typ:
  141. target_cls = cls.TYPES[typ]
  142. if target_cls is not cls:
  143. return target_cls.from_dict(d, app=app)
  144. return Signature(d, app=app)
  145. def __init__(self, task=None, args=None, kwargs=None, options=None,
  146. type=None, subtask_type=None, immutable=False,
  147. app=None, **ex):
  148. self._app = app
  149. if isinstance(task, dict):
  150. super(Signature, self).__init__(task) # works like dict(d)
  151. else:
  152. # Also supports using task class/instance instead of string name.
  153. try:
  154. task_name = task.name
  155. except AttributeError:
  156. task_name = task
  157. else:
  158. self._type = task
  159. super(Signature, self).__init__(
  160. task=task_name, args=tuple(args or ()),
  161. kwargs=kwargs or {},
  162. options=dict(options or {}, **ex),
  163. subtask_type=subtask_type,
  164. immutable=immutable,
  165. chord_size=None,
  166. )
  167. def __call__(self, *partial_args, **partial_kwargs):
  168. """Call the task directly (in the current process)."""
  169. args, kwargs, _ = self._merge(partial_args, partial_kwargs, None)
  170. return self.type(*args, **kwargs)
  171. def delay(self, *partial_args, **partial_kwargs):
  172. """Shortcut to :meth:`apply_async` using star arguments."""
  173. return self.apply_async(partial_args, partial_kwargs)
  174. def apply(self, args=(), kwargs={}, **options):
  175. """Call task locally.
  176. Same as :meth:`apply_async` but executed the task inline instead
  177. of sending a task message.
  178. """
  179. # For callbacks: extra args are prepended to the stored args.
  180. args, kwargs, options = self._merge(args, kwargs, options)
  181. return self.type.apply(args, kwargs, **options)
  182. def apply_async(self, args=(), kwargs={}, route_name=None, **options):
  183. """Apply this task asynchronously.
  184. Arguments:
  185. args (Tuple): Partial args to be prepended to the existing args.
  186. kwargs (Dict): Partial kwargs to be merged with existing kwargs.
  187. options (Dict): Partial options to be merged
  188. with existing options.
  189. Returns:
  190. ~@AsyncResult: promise of future evaluation.
  191. See also:
  192. :meth:`~@Task.apply_async` and the :ref:`guide-calling` guide.
  193. """
  194. try:
  195. _apply = self._apply_async
  196. except IndexError: # pragma: no cover
  197. # no tasks for chain, etc to find type
  198. return
  199. # For callbacks: extra args are prepended to the stored args.
  200. if args or kwargs or options:
  201. args, kwargs, options = self._merge(args, kwargs, options)
  202. else:
  203. args, kwargs, options = self.args, self.kwargs, self.options
  204. # pylint: disable=too-many-function-args
  205. # Borks on this, as it's a property
  206. return _apply(args, kwargs, **options)
  207. def _merge(self, args=(), kwargs={}, options={}, force=False):
  208. if self.immutable and not force:
  209. return (self.args, self.kwargs,
  210. dict(self.options, **options) if options else self.options)
  211. return (tuple(args) + tuple(self.args) if args else self.args,
  212. dict(self.kwargs, **kwargs) if kwargs else self.kwargs,
  213. dict(self.options, **options) if options else self.options)
  214. def clone(self, args=(), kwargs={}, **opts):
  215. """Create a copy of this signature.
  216. Arguments:
  217. args (Tuple): Partial args to be prepended to the existing args.
  218. kwargs (Dict): Partial kwargs to be merged with existing kwargs.
  219. options (Dict): Partial options to be merged with
  220. existing options.
  221. """
  222. # need to deepcopy options so origins links etc. is not modified.
  223. if args or kwargs or opts:
  224. args, kwargs, opts = self._merge(args, kwargs, opts)
  225. else:
  226. args, kwargs, opts = self.args, self.kwargs, self.options
  227. s = Signature.from_dict({'task': self.task, 'args': tuple(args),
  228. 'kwargs': kwargs, 'options': deepcopy(opts),
  229. 'subtask_type': self.subtask_type,
  230. 'chord_size': self.chord_size,
  231. 'immutable': self.immutable}, app=self._app)
  232. s._type = self._type
  233. return s
  234. partial = clone
  235. def freeze(self, _id=None, group_id=None, chord=None,
  236. root_id=None, parent_id=None):
  237. """Finalize the signature by adding a concrete task id.
  238. The task won't be called and you shouldn't call the signature
  239. twice after freezing it as that'll result in two task messages
  240. using the same task id.
  241. Returns:
  242. ~@AsyncResult: promise of future evaluation.
  243. """
  244. # pylint: disable=redefined-outer-name
  245. # XXX chord is also a class in outer scope.
  246. opts = self.options
  247. try:
  248. tid = opts['task_id']
  249. except KeyError:
  250. tid = opts['task_id'] = _id or uuid()
  251. if root_id:
  252. opts['root_id'] = root_id
  253. if parent_id:
  254. opts['parent_id'] = parent_id
  255. if 'reply_to' not in opts:
  256. opts['reply_to'] = self.app.oid
  257. if group_id:
  258. opts['group_id'] = group_id
  259. if chord:
  260. opts['chord'] = chord
  261. # pylint: disable=too-many-function-args
  262. # Borks on this, as it's a property.
  263. return self.AsyncResult(tid)
  264. _freeze = freeze
  265. def replace(self, args=None, kwargs=None, options=None):
  266. """Replace the args, kwargs or options set for this signature.
  267. These are only replaced if the argument for the section is
  268. not :const:`None`.
  269. """
  270. s = self.clone()
  271. if args is not None:
  272. s.args = args
  273. if kwargs is not None:
  274. s.kwargs = kwargs
  275. if options is not None:
  276. s.options = options
  277. return s
  278. def set(self, immutable=None, **options):
  279. """Set arbitrary execution options (same as ``.options.update(…)``).
  280. Returns:
  281. Signature: This is a chaining method call
  282. (i.e., it will return ``self``).
  283. """
  284. if immutable is not None:
  285. self.set_immutable(immutable)
  286. self.options.update(options)
  287. return self
  288. def set_immutable(self, immutable):
  289. self.immutable = immutable
  290. def set_parent_id(self, parent_id):
  291. self.parent_id = parent_id
  292. def _with_list_option(self, key):
  293. items = self.options.setdefault(key, [])
  294. if not isinstance(items, MutableSequence):
  295. items = self.options[key] = [items]
  296. return items
  297. def append_to_list_option(self, key, value):
  298. items = self._with_list_option(key)
  299. if value not in items:
  300. items.append(value)
  301. return value
  302. def extend_list_option(self, key, value):
  303. items = self._with_list_option(key)
  304. items.extend(maybe_list(value))
  305. def link(self, callback):
  306. """Add callback task to be applied if this task succeeds.
  307. Returns:
  308. Signature: the argument passed, for chaining
  309. or use with :func:`~functools.reduce`.
  310. """
  311. return self.append_to_list_option('link', callback)
  312. def link_error(self, errback):
  313. """Add callback task to be applied on error in task execution.
  314. Returns:
  315. Signature: the argument passed, for chaining
  316. or use with :func:`~functools.reduce`.
  317. """
  318. return self.append_to_list_option('link_error', errback)
  319. def on_error(self, errback):
  320. """Version of :meth:`link_error` that supports chaining.
  321. on_error chains the original signature, not the errback so::
  322. >>> add.s(2, 2).on_error(errback.s()).delay()
  323. calls the ``add`` task, not the ``errback`` task, but the
  324. reverse is true for :meth:`link_error`.
  325. """
  326. self.link_error(errback)
  327. return self
  328. def flatten_links(self):
  329. """Return a recursive list of dependencies.
  330. "unchain" if you will, but with links intact.
  331. """
  332. return list(_chain.from_iterable(_chain(
  333. [[self]],
  334. (link.flatten_links()
  335. for link in maybe_list(self.options.get('link')) or [])
  336. )))
  337. def __or__(self, other):
  338. if isinstance(self, group):
  339. if isinstance(other, group):
  340. # group() | group() -> single group
  341. return group(_chain(self.tasks, other.tasks), app=self.app)
  342. # group() | task -> chord
  343. return chord(self, body=other, app=self._app)
  344. elif isinstance(other, group):
  345. # task | group() -> unroll group with one member
  346. other = maybe_unroll_group(other)
  347. return chain(self, other, app=self._app)
  348. if not isinstance(self, chain) and isinstance(other, chain):
  349. # task | chain -> chain
  350. return chain((self,) + other.tasks, app=self._app)
  351. elif isinstance(other, chain):
  352. # chain | chain -> chain
  353. return chain(*self.tasks + other.tasks, app=self._app)
  354. elif isinstance(self, chord):
  355. sig = self.clone()
  356. sig.body = sig.body | other
  357. return sig
  358. elif isinstance(other, Signature):
  359. if isinstance(self, chain):
  360. # chain | task -> chain
  361. return chain(*self.tasks + (other,), app=self._app)
  362. # task | task -> chain
  363. return chain(self, other, app=self._app)
  364. return NotImplemented
  365. def __deepcopy__(self, memo):
  366. memo[id(self)] = self
  367. return dict(self)
  368. def __invert__(self):
  369. return self.apply_async().get()
  370. def __reduce__(self):
  371. # for serialization, the task type is lazily loaded,
  372. # and not stored in the dict itself.
  373. return signature, (dict(self),)
  374. def __json__(self):
  375. return dict(self)
  376. def reprcall(self, *args, **kwargs):
  377. args, kwargs, _ = self._merge(args, kwargs, {}, force=True)
  378. return reprcall(self['task'], args, kwargs)
  379. def election(self):
  380. type = self.type
  381. app = type.app
  382. tid = self.options.get('task_id') or uuid()
  383. with app.producer_or_acquire(None) as P:
  384. props = type.backend.on_task_call(P, tid)
  385. app.control.election(tid, 'task', self.clone(task_id=tid, **props),
  386. connection=P.connection)
  387. return type.AsyncResult(tid)
  388. def __repr__(self):
  389. return self.reprcall()
  390. if JSON_NEEDS_UNICODE_KEYS: # pragma: no cover
  391. def items(self):
  392. for k, v in dict.items(self):
  393. yield k.decode() if isinstance(k, bytes) else k, v
  394. @property
  395. def name(self):
  396. # for duck typing compatibility with Task.name
  397. return self.task
  398. @cached_property
  399. def type(self):
  400. return self._type or self.app.tasks[self['task']]
  401. @cached_property
  402. def app(self):
  403. return self._app or current_app
  404. @cached_property
  405. def AsyncResult(self):
  406. try:
  407. return self.type.AsyncResult
  408. except KeyError: # task not registered
  409. return self.app.AsyncResult
  410. @cached_property
  411. def _apply_async(self):
  412. try:
  413. return self.type.apply_async
  414. except KeyError:
  415. return _partial(self.app.send_task, self['task'])
  416. id = _getitem_property('options.task_id', 'Task UUID')
  417. parent_id = _getitem_property('options.parent_id', 'Task parent UUID.')
  418. root_id = _getitem_property('options.root_id', 'Task root UUID.')
  419. task = _getitem_property('task', 'Name of task.')
  420. args = _getitem_property('args', 'Positional arguments to task.')
  421. kwargs = _getitem_property('kwargs', 'Keyword arguments to task.')
  422. options = _getitem_property('options', 'Task execution options.')
  423. subtask_type = _getitem_property('subtask_type', 'Type of signature')
  424. chord_size = _getitem_property(
  425. 'chord_size', 'Size of chord (if applicable)')
  426. immutable = _getitem_property(
  427. 'immutable', 'Flag set if no longer accepts new arguments')
  428. @Signature.register_type
  429. @python_2_unicode_compatible
  430. class chain(Signature):
  431. """Chain tasks together.
  432. Each tasks follows one another,
  433. by being applied as a callback of the previous task.
  434. Note:
  435. If called with only one argument, then that argument must
  436. be an iterable of tasks to chain: this allows us
  437. to use generator expressions.
  438. Example:
  439. This is effectively :math:`((2 + 2) + 4)`:
  440. .. code-block:: pycon
  441. >>> res = chain(add.s(2, 2), add.s(4))()
  442. >>> res.get()
  443. 8
  444. Calling a chain will return the result of the last task in the chain.
  445. You can get to the other tasks by following the ``result.parent``'s:
  446. .. code-block:: pycon
  447. >>> res.parent.get()
  448. 4
  449. Using a generator expression:
  450. .. code-block:: pycon
  451. >>> lazy_chain = chain(add.s(i) for i in range(10))
  452. >>> res = lazy_chain(3)
  453. Arguments:
  454. *tasks (Signature): List of task signatures to chain.
  455. If only one argument is passed and that argument is
  456. an iterable, then that'll be used as the list of signatures
  457. to chain instead. This means that you can use a generator
  458. expression.
  459. Returns:
  460. ~celery.chain: A lazy signature that can be called to apply the first
  461. task in the chain. When that task succeeed the next task in the
  462. chain is applied, and so on.
  463. """
  464. tasks = _getitem_property('kwargs.tasks', 'Tasks in chain.')
  465. def __init__(self, *tasks, **options):
  466. tasks = (regen(tasks[0]) if len(tasks) == 1 and is_list(tasks[0])
  467. else tasks)
  468. Signature.__init__(
  469. self, 'celery.chain', (), {'tasks': tasks}, **options
  470. )
  471. self._use_link = options.pop('use_link', None)
  472. self.subtask_type = 'chain'
  473. self._frozen = None
  474. def __call__(self, *args, **kwargs):
  475. if self.tasks:
  476. return self.apply_async(args, kwargs)
  477. def clone(self, *args, **kwargs):
  478. s = Signature.clone(self, *args, **kwargs)
  479. s.kwargs['tasks'] = [sig.clone() for sig in s.kwargs['tasks']]
  480. return s
  481. def apply_async(self, args=(), kwargs={}, **options):
  482. # python is best at unpacking kwargs, so .run is here to do that.
  483. app = self.app
  484. if app.conf.task_always_eager:
  485. return self.apply(args, kwargs, **options)
  486. return self.run(args, kwargs, app=app, **(
  487. dict(self.options, **options) if options else self.options))
  488. def run(self, args=(), kwargs={}, group_id=None, chord=None,
  489. task_id=None, link=None, link_error=None, publisher=None,
  490. producer=None, root_id=None, parent_id=None, app=None, **options):
  491. # pylint: disable=redefined-outer-name
  492. # XXX chord is also a class in outer scope.
  493. app = app or self.app
  494. use_link = self._use_link
  495. if use_link is None and app.conf.task_protocol == 1:
  496. use_link = True
  497. args = (tuple(args) + tuple(self.args)
  498. if args and not self.immutable else self.args)
  499. if self._frozen:
  500. tasks, results = self._frozen
  501. else:
  502. tasks, results = self.prepare_steps(
  503. args, self.tasks, root_id, parent_id, link_error, app,
  504. task_id, group_id, chord,
  505. )
  506. if results:
  507. if link:
  508. tasks[0].extend_list_option('link', link)
  509. first_task = tasks.pop()
  510. first_task.apply_async(
  511. chain=tasks if not use_link else None, **options)
  512. return results[0]
  513. def freeze(self, _id=None, group_id=None, chord=None,
  514. root_id=None, parent_id=None):
  515. # pylint: disable=redefined-outer-name
  516. # XXX chord is also a class in outer scope.
  517. _, results = self._frozen = self.prepare_steps(
  518. self.args, self.tasks, root_id, parent_id, None,
  519. self.app, _id, group_id, chord, clone=False,
  520. )
  521. return results[0]
  522. def prepare_steps(self, args, tasks,
  523. root_id=None, parent_id=None, link_error=None, app=None,
  524. last_task_id=None, group_id=None, chord_body=None,
  525. clone=True, from_dict=Signature.from_dict):
  526. app = app or self.app
  527. # use chain message field for protocol 2 and later.
  528. # this avoids pickle blowing the stack on the recursion
  529. # required by linking task together in a tree structure.
  530. # (why is pickle using recursion? or better yet why cannot python
  531. # do tail call optimization making recursion actually useful?)
  532. use_link = self._use_link
  533. if use_link is None and app.conf.task_protocol == 1:
  534. use_link = True
  535. steps = deque(tasks)
  536. steps_pop = steps.pop
  537. steps_extend = steps.extend
  538. prev_task = None
  539. prev_res = prev_prev_res = None
  540. tasks, results = [], []
  541. i = 0
  542. while steps:
  543. task = steps_pop()
  544. is_first_task, is_last_task = not steps, not i
  545. if not isinstance(task, abstract.CallableSignature):
  546. task = from_dict(task, app=app)
  547. if isinstance(task, group):
  548. task = maybe_unroll_group(task)
  549. # first task gets partial args from chain
  550. if clone:
  551. task = task.clone(args) if is_first_task else task.clone()
  552. elif is_first_task:
  553. task.args = tuple(args) + tuple(task.args)
  554. if isinstance(task, chain):
  555. # splice the chain
  556. steps_extend(task.tasks)
  557. continue
  558. if isinstance(task, group) and prev_task:
  559. # automatically upgrade group(...) | s to chord(group, s)
  560. # for chords we freeze by pretending it's a normal
  561. # signature instead of a group.
  562. tasks.pop()
  563. results.pop()
  564. task = chord(
  565. task, body=prev_task,
  566. task_id=prev_res.task_id, root_id=root_id, app=app,
  567. )
  568. prev_res = prev_prev_res
  569. if is_last_task:
  570. # chain(task_id=id) means task id is set for the last task
  571. # in the chain. If the chord is part of a chord/group
  572. # then that chord/group must synchronize based on the
  573. # last task in the chain, so we only set the group_id and
  574. # chord callback for the last task.
  575. res = task.freeze(
  576. last_task_id,
  577. root_id=root_id, group_id=group_id, chord=chord_body,
  578. )
  579. else:
  580. res = task.freeze(root_id=root_id)
  581. i += 1
  582. if prev_task:
  583. prev_task.set_parent_id(task.id)
  584. if use_link:
  585. # link previous task to this task.
  586. task.link(prev_task)
  587. if prev_res:
  588. prev_res.parent = res
  589. if is_first_task and parent_id is not None:
  590. task.set_parent_id(parent_id)
  591. if link_error:
  592. for errback in maybe_list(link_error):
  593. task.link_error(errback)
  594. tasks.append(task)
  595. results.append(res)
  596. prev_task, prev_prev_res, prev_res = (
  597. task, prev_res, res,
  598. )
  599. if root_id is None and tasks:
  600. root_id = tasks[-1].id
  601. for task in reversed(tasks):
  602. task.options['root_id'] = root_id
  603. return tasks, results
  604. def apply(self, args=(), kwargs={}, **options):
  605. last, fargs = None, args
  606. for task in self.tasks:
  607. res = task.clone(fargs).apply(
  608. last and (last.get(),), **dict(self.options, **options))
  609. res.parent, last, fargs = last, res, None
  610. return last
  611. @classmethod
  612. def from_dict(cls, d, app=None):
  613. tasks = d['kwargs']['tasks']
  614. if tasks:
  615. if isinstance(tasks, tuple): # aaaargh
  616. tasks = d['kwargs']['tasks'] = list(tasks)
  617. # First task must be signature object to get app
  618. tasks[0] = maybe_signature(tasks[0], app=app)
  619. return _upgrade(d, chain(tasks, app=app, **d['options']))
  620. @property
  621. def app(self):
  622. app = self._app
  623. if app is None:
  624. try:
  625. app = self.tasks[0]._app
  626. except LookupError:
  627. pass
  628. return app or current_app
  629. def __repr__(self):
  630. if not self.tasks:
  631. return '<{0}@{1:#x}: empty>'.format(
  632. type(self).__name__, id(self))
  633. return ' | '.join(repr(t) for t in self.tasks)
  634. class _basemap(Signature):
  635. _task_name = None
  636. _unpack_args = itemgetter('task', 'it')
  637. def __init__(self, task, it, **options):
  638. Signature.__init__(
  639. self, self._task_name, (),
  640. {'task': task, 'it': regen(it)}, immutable=True, **options
  641. )
  642. def apply_async(self, args=(), kwargs={}, **opts):
  643. # need to evaluate generators
  644. task, it = self._unpack_args(self.kwargs)
  645. return self.type.apply_async(
  646. (), {'task': task, 'it': list(it)},
  647. route_name=task_name_from(self.kwargs.get('task')), **opts
  648. )
  649. @classmethod
  650. def from_dict(cls, d, app=None):
  651. return _upgrade(
  652. d, cls(*cls._unpack_args(d['kwargs']), app=app, **d['options']),
  653. )
  654. @Signature.register_type
  655. @python_2_unicode_compatible
  656. class xmap(_basemap):
  657. """Map operation for tasks.
  658. Note:
  659. Tasks executed sequentially in process, this is not a
  660. parallel operation like :class:`group`.
  661. """
  662. _task_name = 'celery.map'
  663. def __repr__(self):
  664. task, it = self._unpack_args(self.kwargs)
  665. return '[{0}(x) for x in {1}]'.format(task.task,
  666. truncate(repr(it), 100))
  667. @Signature.register_type
  668. @python_2_unicode_compatible
  669. class xstarmap(_basemap):
  670. """Map operation for tasks, using star arguments."""
  671. _task_name = 'celery.starmap'
  672. def __repr__(self):
  673. task, it = self._unpack_args(self.kwargs)
  674. return '[{0}(*x) for x in {1}]'.format(task.task,
  675. truncate(repr(it), 100))
  676. @Signature.register_type
  677. class chunks(Signature):
  678. """Partition of tasks in n chunks."""
  679. _unpack_args = itemgetter('task', 'it', 'n')
  680. def __init__(self, task, it, n, **options):
  681. Signature.__init__(
  682. self, 'celery.chunks', (),
  683. {'task': task, 'it': regen(it), 'n': n},
  684. immutable=True, **options
  685. )
  686. @classmethod
  687. def from_dict(cls, d, app=None):
  688. return _upgrade(
  689. d, chunks(*cls._unpack_args(
  690. d['kwargs']), app=app, **d['options']),
  691. )
  692. def apply_async(self, args=(), kwargs={}, **opts):
  693. return self.group().apply_async(
  694. args, kwargs,
  695. route_name=task_name_from(self.kwargs.get('task')), **opts
  696. )
  697. def __call__(self, **options):
  698. return self.apply_async(**options)
  699. def group(self):
  700. # need to evaluate generators
  701. task, it, n = self._unpack_args(self.kwargs)
  702. return group((xstarmap(task, part, app=self._app)
  703. for part in _chunks(iter(it), n)),
  704. app=self._app)
  705. @classmethod
  706. def apply_chunks(cls, task, it, n, app=None):
  707. return cls(task, it, n, app=app)()
  708. def _maybe_group(tasks, app):
  709. if isinstance(tasks, dict):
  710. tasks = signature(tasks, app=app)
  711. if isinstance(tasks, (group, chain)):
  712. tasks = tasks.tasks
  713. elif isinstance(tasks, abstract.CallableSignature):
  714. tasks = [tasks]
  715. else:
  716. tasks = [signature(t, app=app) for t in tasks]
  717. return tasks
  718. @Signature.register_type
  719. @python_2_unicode_compatible
  720. class group(Signature):
  721. """Creates a group of tasks to be executed in parallel.
  722. A group is lazy so you must call it to take action and evaluate
  723. the group.
  724. Note:
  725. If only one argument is passed, and that argument is an iterable
  726. then that'll be used as the list of tasks instead: this
  727. allows us to use ``group`` with generator expressions.
  728. Example:
  729. >>> lazy_group = group([add.s(2, 2), add.s(4, 4)])
  730. >>> promise = lazy_group() # <-- evaluate: returns lazy result.
  731. >>> promise.get() # <-- will wait for the task to return
  732. [4, 8]
  733. Arguments:
  734. *tasks (Signature): A list of signatures that this group will call.
  735. If there's only one argument, and that argument is an iterable,
  736. then that'll define the list of signatures instead.
  737. **options (Any): Execution options applied to all tasks
  738. in the group.
  739. Returns:
  740. ~celery.group: signature that when called will then call all of the
  741. tasks in the group (and return a :class:`GroupResult` instance
  742. that can be used to inspect the state of the group).
  743. """
  744. tasks = _getitem_property('kwargs.tasks', 'Tasks in group.')
  745. def __init__(self, *tasks, **options):
  746. if len(tasks) == 1:
  747. tasks = tasks[0]
  748. if isinstance(tasks, group):
  749. tasks = tasks.tasks
  750. if not isinstance(tasks, _regen):
  751. tasks = regen(tasks)
  752. Signature.__init__(
  753. self, 'celery.group', (), {'tasks': tasks}, **options
  754. )
  755. self.subtask_type = 'group'
  756. @classmethod
  757. def from_dict(cls, d, app=None):
  758. return _upgrade(
  759. d, group(d['kwargs']['tasks'], app=app, **d['options']),
  760. )
  761. def __len__(self):
  762. return len(self.tasks)
  763. def _prepared(self, tasks, partial_args, group_id, root_id, app,
  764. CallableSignature=abstract.CallableSignature,
  765. from_dict=Signature.from_dict,
  766. isinstance=isinstance, tuple=tuple):
  767. for task in tasks:
  768. if isinstance(task, CallableSignature):
  769. # local sigs are always of type Signature, and we
  770. # clone them to make sure we don't modify the originals.
  771. task = task.clone()
  772. else:
  773. # serialized sigs must be converted to Signature.
  774. task = from_dict(task, app=app)
  775. if isinstance(task, group):
  776. # needs yield_from :(
  777. unroll = task._prepared(
  778. task.tasks, partial_args, group_id, root_id, app,
  779. )
  780. for taskN, resN in unroll:
  781. yield taskN, resN
  782. else:
  783. if partial_args and not task.immutable:
  784. task.args = tuple(partial_args) + tuple(task.args)
  785. yield task, task.freeze(group_id=group_id, root_id=root_id)
  786. def _apply_tasks(self, tasks, producer=None, app=None, p=None,
  787. add_to_parent=None, chord=None,
  788. args=None, kwargs=None, **options):
  789. # pylint: disable=redefined-outer-name
  790. # XXX chord is also a class in outer scope.
  791. app = app or self.app
  792. with app.producer_or_acquire(producer) as producer:
  793. for sig, res in tasks:
  794. sig.apply_async(producer=producer, add_to_parent=False,
  795. chord=sig.options.get('chord') or chord,
  796. args=args, kwargs=kwargs,
  797. **options)
  798. # adding callback to result, such that it will gradually
  799. # fulfill the barrier.
  800. #
  801. # Using barrier.add would use result.then, but we need
  802. # to add the weak argument here to only create a weak
  803. # reference to the object.
  804. if p and not p.cancelled and not p.ready:
  805. p.size += 1
  806. res.then(p, weak=True)
  807. yield res # <-- r.parent, etc set in the frozen result.
  808. def _freeze_gid(self, options):
  809. # remove task_id and use that as the group_id,
  810. # if we don't remove it then every task will have the same id...
  811. options = dict(self.options, **options)
  812. options['group_id'] = group_id = (
  813. options.pop('task_id', uuid()))
  814. return options, group_id, options.get('root_id')
  815. def set_parent_id(self, parent_id):
  816. for task in self.tasks:
  817. task.set_parent_id(parent_id)
  818. def apply_async(self, args=(), kwargs=None, add_to_parent=True,
  819. producer=None, **options):
  820. app = self.app
  821. if app.conf.task_always_eager:
  822. return self.apply(args, kwargs, **options)
  823. if not self.tasks:
  824. return self.freeze()
  825. options, group_id, root_id = self._freeze_gid(options)
  826. tasks = self._prepared(self.tasks, [], group_id, root_id, app)
  827. p = barrier()
  828. results = list(self._apply_tasks(tasks, producer, app, p,
  829. args=args, kwargs=kwargs, **options))
  830. result = self.app.GroupResult(group_id, results, ready_barrier=p)
  831. p.finalize()
  832. # - Special case of group(A.s() | group(B.s(), C.s()))
  833. # That is, group with single item that's a chain but the
  834. # last task in that chain is a group.
  835. #
  836. # We cannot actually support arbitrary GroupResults in chains,
  837. # but this special case we can.
  838. if len(result) == 1 and isinstance(result[0], GroupResult):
  839. result = result[0]
  840. parent_task = app.current_worker_task
  841. if add_to_parent and parent_task:
  842. parent_task.add_trail(result)
  843. return result
  844. def apply(self, args=(), kwargs={}, **options):
  845. app = self.app
  846. if not self.tasks:
  847. return self.freeze() # empty group returns GroupResult
  848. options, group_id, root_id = self._freeze_gid(options)
  849. tasks = self._prepared(self.tasks, [], group_id, root_id, app)
  850. return app.GroupResult(group_id, [
  851. sig.apply(args=args, kwargs=kwargs, **options) for sig, _ in tasks
  852. ])
  853. def set_immutable(self, immutable):
  854. for task in self.tasks:
  855. task.set_immutable(immutable)
  856. def link(self, sig):
  857. # Simply link to first task
  858. sig = sig.clone().set(immutable=True)
  859. return self.tasks[0].link(sig)
  860. def link_error(self, sig):
  861. sig = sig.clone().set(immutable=True)
  862. return self.tasks[0].link_error(sig)
  863. def __call__(self, *partial_args, **options):
  864. return self.apply_async(partial_args, **options)
  865. def _freeze_unroll(self, new_tasks, group_id, chord, root_id, parent_id):
  866. # pylint: disable=redefined-outer-name
  867. # XXX chord is also a class in outer scope.
  868. stack = deque(self.tasks)
  869. while stack:
  870. task = maybe_signature(stack.popleft(), app=self._app).clone()
  871. if isinstance(task, group):
  872. stack.extendleft(task.tasks)
  873. else:
  874. new_tasks.append(task)
  875. yield task.freeze(group_id=group_id,
  876. chord=chord, root_id=root_id,
  877. parent_id=parent_id)
  878. def freeze(self, _id=None, group_id=None, chord=None,
  879. root_id=None, parent_id=None):
  880. # pylint: disable=redefined-outer-name
  881. # XXX chord is also a class in outer scope.
  882. opts = self.options
  883. try:
  884. gid = opts['task_id']
  885. except KeyError:
  886. gid = opts['task_id'] = uuid()
  887. if group_id:
  888. opts['group_id'] = group_id
  889. if chord:
  890. opts['chord'] = chord
  891. root_id = opts.setdefault('root_id', root_id)
  892. parent_id = opts.setdefault('parent_id', parent_id)
  893. new_tasks = []
  894. # Need to unroll subgroups early so that chord gets the
  895. # right result instance for chord_unlock etc.
  896. results = list(self._freeze_unroll(
  897. new_tasks, group_id, chord, root_id, parent_id,
  898. ))
  899. if isinstance(self.tasks, MutableSequence):
  900. self.tasks[:] = new_tasks
  901. else:
  902. self.tasks = new_tasks
  903. return self.app.GroupResult(gid, results)
  904. _freeze = freeze
  905. def skew(self, start=1.0, stop=None, step=1.0):
  906. it = fxrange(start, stop, step, repeatlast=True)
  907. for task in self.tasks:
  908. task.set(countdown=next(it))
  909. return self
  910. def __iter__(self):
  911. return iter(self.tasks)
  912. def __repr__(self):
  913. return 'group({0.tasks!r})'.format(self)
  914. @property
  915. def app(self):
  916. app = self._app
  917. if app is None:
  918. try:
  919. app = self.tasks[0].app
  920. except LookupError:
  921. pass
  922. return app if app is not None else current_app
  923. @Signature.register_type
  924. @python_2_unicode_compatible
  925. class chord(Signature):
  926. r"""Barrier synchronization primitive.
  927. A chord consists of a header and a body.
  928. The header is a group of tasks that must complete before the callback is
  929. called. A chord is essentially a callback for a group of tasks.
  930. The body is applied with the return values of all the header
  931. tasks as a list.
  932. Example:
  933. The chord:
  934. .. code-block:: pycon
  935. >>> res = chord([add.s(2, 2), add.s(4, 4)])(sum_task.s())
  936. is effectively :math:`\Sigma ((2 + 2) + (4 + 4))`:
  937. .. code-block:: pycon
  938. >>> res.get()
  939. 12
  940. """
  941. def __init__(self, header, body=None, task='celery.chord',
  942. args=(), kwargs={}, app=None, **options):
  943. Signature.__init__(
  944. self, task, args,
  945. dict(kwargs=kwargs, header=_maybe_group(header, app),
  946. body=maybe_signature(body, app=app)), app=app, **options
  947. )
  948. self.subtask_type = 'chord'
  949. def freeze(self, _id=None, group_id=None, chord=None,
  950. root_id=None, parent_id=None):
  951. # pylint: disable=redefined-outer-name
  952. # XXX chord is also a class in outer scope.
  953. if not isinstance(self.tasks, group):
  954. self.tasks = group(self.tasks, app=self.app)
  955. bodyres = self.body.freeze(_id, parent_id=self.id, root_id=root_id)
  956. self.tasks.freeze(
  957. parent_id=parent_id, root_id=root_id, chord=self.body)
  958. self.id = self.tasks.id
  959. self.body.set_parent_id(self.id)
  960. return bodyres
  961. def set_parent_id(self, parent_id):
  962. tasks = self.tasks
  963. if isinstance(tasks, group):
  964. tasks = tasks.tasks
  965. for task in tasks:
  966. task.set_parent_id(parent_id)
  967. self.parent_id = parent_id
  968. @classmethod
  969. def from_dict(cls, d, app=None):
  970. args, d['kwargs'] = cls._unpack_args(**d['kwargs'])
  971. return _upgrade(d, cls(*args, app=app, **d))
  972. @staticmethod
  973. def _unpack_args(header=None, body=None, **kwargs):
  974. # Python signatures are better at extracting keys from dicts
  975. # than manually popping things off.
  976. return (header, body), kwargs
  977. @cached_property
  978. def app(self):
  979. return self._get_app(self.body)
  980. def _get_app(self, body=None):
  981. app = self._app
  982. if app is None:
  983. try:
  984. tasks = self.tasks.tasks # is a group
  985. except AttributeError:
  986. tasks = self.tasks
  987. app = tasks[0]._app
  988. if app is None and body is not None:
  989. app = body._app
  990. return app if app is not None else current_app
  991. def apply_async(self, args=(), kwargs={}, task_id=None,
  992. producer=None, publisher=None, connection=None,
  993. router=None, result_cls=None, **options):
  994. kwargs = kwargs or {}
  995. args = (tuple(args) + tuple(self.args)
  996. if args and not self.immutable else self.args)
  997. body = kwargs.pop('body', None) or self.kwargs['body']
  998. kwargs = dict(self.kwargs['kwargs'], **kwargs)
  999. body = body.clone(**options)
  1000. app = self._get_app(body)
  1001. tasks = (self.tasks.clone() if isinstance(self.tasks, group)
  1002. else group(self.tasks, app=app))
  1003. if app.conf.task_always_eager:
  1004. return self.apply(args, kwargs,
  1005. body=body, task_id=task_id, **options)
  1006. return self.run(tasks, body, args, task_id=task_id, **options)
  1007. def apply(self, args=(), kwargs={}, propagate=True, body=None, **options):
  1008. body = self.body if body is None else body
  1009. tasks = (self.tasks.clone() if isinstance(self.tasks, group)
  1010. else group(self.tasks, app=self.app))
  1011. return body.apply(
  1012. args=(tasks.apply(args, kwargs).get(propagate=propagate),),
  1013. )
  1014. def _traverse_tasks(self, tasks, value=None):
  1015. stack = deque(list(tasks))
  1016. while stack:
  1017. task = stack.popleft()
  1018. if isinstance(task, group):
  1019. stack.extend(task.tasks)
  1020. else:
  1021. yield task if value is None else value
  1022. def __length_hint__(self):
  1023. return sum(self._traverse_tasks(self.tasks, 1))
  1024. def run(self, header, body, partial_args, app=None, interval=None,
  1025. countdown=1, max_retries=None, eager=False,
  1026. task_id=None, **options):
  1027. app = app or self._get_app(body)
  1028. group_id = uuid()
  1029. root_id = body.options.get('root_id')
  1030. body.chord_size = self.__length_hint__()
  1031. options = dict(self.options, **options) if options else self.options
  1032. if options:
  1033. options.pop('task_id', None)
  1034. body.options.update(options)
  1035. results = header.freeze(
  1036. group_id=group_id, chord=body, root_id=root_id).results
  1037. bodyres = body.freeze(task_id, root_id=root_id)
  1038. parent = app.backend.apply_chord(
  1039. header, partial_args, group_id, body,
  1040. interval=interval, countdown=countdown,
  1041. options=options, max_retries=max_retries,
  1042. result=results)
  1043. bodyres.parent = parent
  1044. return bodyres
  1045. def __call__(self, body=None, **options):
  1046. return self.apply_async((), {'body': body} if body else {}, **options)
  1047. def clone(self, *args, **kwargs):
  1048. s = Signature.clone(self, *args, **kwargs)
  1049. # need to make copy of body
  1050. try:
  1051. s.kwargs['body'] = s.kwargs['body'].clone()
  1052. except (AttributeError, KeyError):
  1053. pass
  1054. return s
  1055. def link(self, callback):
  1056. self.body.link(callback)
  1057. return callback
  1058. def link_error(self, errback):
  1059. self.body.link_error(errback)
  1060. return errback
  1061. def set_immutable(self, immutable):
  1062. # changes mutability of header only, not callback.
  1063. for task in self.tasks:
  1064. task.set_immutable(immutable)
  1065. def __repr__(self):
  1066. if self.body:
  1067. return self.body.reprcall(self.tasks)
  1068. return '<chord without body: {0.tasks!r}>'.format(self)
  1069. tasks = _getitem_property('kwargs.header', 'Tasks in chord header.')
  1070. body = _getitem_property('kwargs.body', 'Body task of chord.')
  1071. def signature(varies, *args, **kwargs):
  1072. """Create new signature.
  1073. - if the first argument is a signature already then it's cloned.
  1074. - if the first argument is a dict, then a Signature version is returned.
  1075. Returns:
  1076. Signature: The resulting signature.
  1077. """
  1078. app = kwargs.get('app')
  1079. if isinstance(varies, dict):
  1080. if isinstance(varies, abstract.CallableSignature):
  1081. return varies.clone()
  1082. return Signature.from_dict(varies, app=app)
  1083. return Signature(varies, *args, **kwargs)
  1084. subtask = signature # XXX compat
  1085. def maybe_signature(d, app=None):
  1086. """Ensure obj is a signature, or None."""
  1087. if d is not None:
  1088. if (isinstance(d, dict) and
  1089. not isinstance(d, abstract.CallableSignature)):
  1090. d = signature(d)
  1091. if app is not None:
  1092. d._app = app
  1093. return d
  1094. maybe_subtask = maybe_signature # XXX compat