|
8 | 8 | package org.elasticsearch.compute.operator.exchange; |
9 | 9 |
|
10 | 10 | import org.elasticsearch.action.ActionListener; |
11 | | -import org.elasticsearch.action.ActionRunnable; |
12 | | -import org.elasticsearch.action.support.RefCountingRunnable; |
13 | 11 | import org.elasticsearch.action.support.SubscribableListener; |
14 | 12 | import org.elasticsearch.common.util.concurrent.AbstractRunnable; |
15 | 13 | import org.elasticsearch.common.util.concurrent.ConcurrentCollections; |
|
19 | 17 | import org.elasticsearch.core.Releasable; |
20 | 18 | import org.elasticsearch.tasks.TaskCancelledException; |
21 | 19 |
|
22 | | -import java.util.List; |
23 | 20 | import java.util.Map; |
24 | 21 | import java.util.concurrent.Executor; |
25 | 22 | import java.util.concurrent.atomic.AtomicInteger; |
@@ -51,28 +48,12 @@ public final class ExchangeSourceHandler { |
51 | 48 | * @param maxBufferSize the maximum size of the exchange buffer. A larger buffer reduces ``pauses`` but uses more memory, |
52 | 49 | * which could otherwise be allocated for other purposes. |
53 | 50 | * @param fetchExecutor the executor used to fetch pages. |
54 | | - * @param completionListener a listener that will be notified when the exchange source handler completes |
55 | 51 | */ |
56 | | - public ExchangeSourceHandler(int maxBufferSize, Executor fetchExecutor, ActionListener<Void> completionListener) { |
| 52 | + public ExchangeSourceHandler(int maxBufferSize, Executor fetchExecutor) { |
57 | 53 | this.buffer = new ExchangeBuffer(maxBufferSize); |
58 | 54 | this.fetchExecutor = fetchExecutor; |
59 | 55 | this.outstandingSinks = new PendingInstances(() -> buffer.finish(false)); |
60 | | - final PendingInstances closingSinks = new PendingInstances(() -> {}); |
61 | | - closingSinks.trackNewInstance(); |
62 | | - this.outstandingSources = new PendingInstances(() -> finishEarly(true, ActionListener.running(closingSinks::finishInstance))); |
63 | | - buffer.addCompletionListener(ActionListener.running(() -> { |
64 | | - final ActionListener<Void> listener = ActionListener.assertAtLeastOnce(completionListener); |
65 | | - try (RefCountingRunnable refs = new RefCountingRunnable(ActionRunnable.run(listener, this::checkFailure))) { |
66 | | - closingSinks.completion.addListener(refs.acquireListener()); |
67 | | - for (PendingInstances pending : List.of(outstandingSinks, outstandingSources)) { |
68 | | - // Create an outstanding instance and then finish to complete the completionListener |
69 | | - // if we haven't registered any instances of exchange sinks or exchange sources before. |
70 | | - pending.trackNewInstance(); |
71 | | - pending.completion.addListener(refs.acquireListener()); |
72 | | - pending.finishInstance(); |
73 | | - } |
74 | | - } |
75 | | - })); |
| 56 | + this.outstandingSources = new PendingInstances(() -> finishEarly(true, ActionListener.noop())); |
76 | 57 | } |
77 | 58 |
|
78 | 59 | private void checkFailure() { |
@@ -271,7 +252,13 @@ public void addRemoteSink( |
271 | 252 | final ActionListener<Void> sinkListener = ActionListener.assertAtLeastOnce( |
272 | 253 | ActionListener.notifyOnce(ActionListener.runBefore(listener, () -> remoteSinks.remove(sinkId))) |
273 | 254 | ); |
| 255 | + final Releasable emptySink = addEmptySink(); |
274 | 256 | fetchExecutor.execute(new AbstractRunnable() { |
| 257 | + @Override |
| 258 | + public void onAfter() { |
| 259 | + emptySink.close(); |
| 260 | + } |
| 261 | + |
275 | 262 | @Override |
276 | 263 | public void onFailure(Exception e) { |
277 | 264 | if (failFast) { |
|
0 commit comments