forked from dask/distributed
/
test_as_completed.py
256 lines (192 loc) · 6.21 KB
/
test_as_completed.py
1
2
3
4
5
6
7
8
9
10
11
12
13
14
15
16
17
18
19
20
21
22
23
24
25
26
27
28
29
30
31
32
33
34
35
36
37
38
39
40
41
42
43
44
45
46
47
48
49
50
51
52
53
54
55
56
57
58
59
60
61
62
63
64
65
66
67
68
69
70
71
72
73
74
75
76
77
78
79
80
81
82
83
84
85
86
87
88
89
90
91
92
93
94
95
96
97
98
99
100
101
102
103
104
105
106
107
108
109
110
111
112
113
114
115
116
117
118
119
120
121
122
123
124
125
126
127
128
129
130
131
132
133
134
135
136
137
138
139
140
141
142
143
144
145
146
147
148
149
150
151
152
153
154
155
156
157
158
159
160
161
162
163
164
165
166
167
168
169
170
171
172
173
174
175
176
177
178
179
180
181
182
183
184
185
186
187
188
189
190
191
192
193
194
195
196
197
198
199
200
201
202
203
204
205
206
207
208
209
210
211
212
213
214
215
216
217
218
219
220
221
222
223
224
225
226
227
228
229
230
231
232
233
234
235
236
237
238
239
240
241
242
243
244
245
246
247
248
249
250
251
252
253
254
255
256
from concurrent.futures import CancelledError
from collections.abc import Iterator
from operator import add
import queue
import random
from time import sleep
import pytest
from tornado import gen
from distributed.client import _as_completed, as_completed, _first_completed
from distributed.utils_test import gen_cluster, inc, throws
from distributed.utils_test import client, cluster_fixture, loop # noqa: F401
@gen_cluster(client=True)
def test__as_completed(c, s, a, b):
x = c.submit(inc, 1)
y = c.submit(inc, 1)
z = c.submit(inc, 2)
q = queue.Queue()
yield _as_completed([x, y, z], q)
assert q.qsize() == 3
assert {q.get(), q.get(), q.get()} == {x, y, z}
result = yield _first_completed([x, y, z])
assert result in [x, y, z]
def test_as_completed(client):
x = client.submit(inc, 1)
y = client.submit(inc, 2)
z = client.submit(inc, 1)
seq = as_completed([x, y, z])
assert seq.count() == 3
assert isinstance(seq, Iterator)
assert set(seq) == {x, y, z}
assert seq.count() == 0
assert list(as_completed([])) == []
def test_as_completed_with_non_futures(client):
with pytest.raises(TypeError):
list(as_completed([1, 2, 3]))
def test_as_completed_add(client):
total = 0
expected = sum(map(inc, range(10)))
futures = client.map(inc, range(10))
ac = as_completed(futures)
for future in ac:
result = future.result()
total += result
if random.random() < 0.5:
future = client.submit(add, future, 10)
ac.add(future)
expected += result + 10
assert total == expected
def test_as_completed_update(client):
total = 0
todo = list(range(10))
expected = sum(map(inc, todo))
ac = as_completed([])
while todo or not ac.is_empty():
if todo:
work, todo = todo[:4], todo[4:]
ac.update(client.map(inc, work))
batch = ac.next_batch(block=True)
total += sum(r.result() for r in batch)
assert total == expected
def test_as_completed_repeats(client):
ac = as_completed()
x = client.submit(inc, 1)
ac.add(x)
ac.add(x)
assert next(ac) is x
assert next(ac) is x
with pytest.raises(StopIteration):
next(ac)
ac.add(x)
assert next(ac) is x
def test_as_completed_is_empty(client):
ac = as_completed()
assert ac.is_empty()
x = client.submit(inc, 1)
ac.add(x)
assert not ac.is_empty()
assert next(ac) is x
assert ac.is_empty()
def test_as_completed_cancel(client):
x = client.submit(inc, 1)
y = client.submit(inc, 1)
ac = as_completed([x, y])
x.cancel()
assert next(ac) is x or y
assert next(ac) is y or x
with pytest.raises(queue.Empty):
ac.queue.get(timeout=0.1)
res = list(as_completed([x, y, x]))
assert len(res) == 3
assert set(res) == {x, y}
assert res.count(x) == 2
def test_as_completed_cancel_last(client):
w = client.submit(inc, 0.3)
x = client.submit(inc, 1)
y = client.submit(inc, 0.3)
@gen.coroutine
def _():
yield gen.sleep(0.1)
yield w.cancel(asynchronous=True)
yield y.cancel(asynchronous=True)
client.loop.add_callback(_)
ac = as_completed([x, y])
result = set(ac)
assert result == {x, y}
@gen_cluster(client=True)
def test_async_for_py2_equivalent(c, s, a, b):
futures = c.map(sleep, [0.01] * 3, pure=False)
seq = as_completed(futures)
x = yield seq.__anext__()
y = yield seq.__anext__()
z = yield seq.__anext__()
assert x.done()
assert y.done()
assert z.done()
assert x.key != y.key
with pytest.raises(StopAsyncIteration):
yield seq.__anext__()
@gen_cluster(client=True)
def test_as_completed_error_async(c, s, a, b):
x = c.submit(throws, 1)
y = c.submit(inc, 1)
ac = as_completed([x, y])
first = yield ac.__anext__()
second = yield ac.__anext__()
result = {first, second}
assert result == {x, y}
assert x.status == "error"
assert y.status == "finished"
def test_as_completed_error(client):
x = client.submit(throws, 1)
y = client.submit(inc, 1)
ac = as_completed([x, y])
result = set(ac)
assert result == {x, y}
assert x.status == "error"
assert y.status == "finished"
def test_as_completed_with_results(client):
x = client.submit(throws, 1)
y = client.submit(inc, 5)
z = client.submit(inc, 1)
ac = as_completed([x, y, z], with_results=True)
y.cancel()
with pytest.raises(RuntimeError) as exc:
res = list(ac)
assert str(exc.value) == "hello!"
@gen_cluster(client=True)
def test_as_completed_with_results_async(c, s, a, b):
x = c.submit(throws, 1)
y = c.submit(inc, 5)
z = c.submit(inc, 1)
ac = as_completed([x, y, z], with_results=True)
yield y.cancel()
with pytest.raises(RuntimeError) as exc:
first = yield ac.__anext__()
second = yield ac.__anext__()
third = yield ac.__anext__()
assert str(exc.value) == "hello!"
def test_as_completed_with_results_no_raise(client):
x = client.submit(throws, 1)
y = client.submit(inc, 5)
z = client.submit(inc, 1)
ac = as_completed([x, y, z], with_results=True, raise_errors=False)
y.cancel()
res = list(ac)
dd = {r[0]: r[1:] for r in res}
assert set(dd.keys()) == {y, x, z}
assert x.status == "error"
assert y.status == "cancelled"
assert z.status == "finished"
assert isinstance(dd[y][0], CancelledError) or dd[y][0] == 6
assert isinstance(dd[x][0][1], RuntimeError)
assert dd[z][0] == 2
@gen_cluster(client=True)
def test_as_completed_with_results_no_raise_async(c, s, a, b):
x = c.submit(throws, 1)
y = c.submit(inc, 5)
z = c.submit(inc, 1)
ac = as_completed([x, y, z], with_results=True, raise_errors=False)
c.loop.add_callback(y.cancel)
first = yield ac.__anext__()
second = yield ac.__anext__()
third = yield ac.__anext__()
res = [first, second, third]
dd = {r[0]: r[1:] for r in res}
assert set(dd.keys()) == {y, x, z}
assert x.status == "error"
assert y.status == "cancelled"
assert z.status == "finished"
assert isinstance(dd[y][0], CancelledError)
assert isinstance(dd[x][0][1], RuntimeError)
assert dd[z][0] == 2