Skip to content

Commit f79fe67

Browse files
authored
fix: DH-18482: EventDrivenUpdateGraph: Allow two concurrent threads to safely requestRefresh(), and ensure clean errors for nested refresh attempts (#6603)
1 parent 24f1d83 commit f79fe67

File tree

3 files changed

+77
-9
lines changed

3 files changed

+77
-9
lines changed

engine/table/src/main/java/io/deephaven/engine/updategraph/impl/BaseUpdateGraph.java

Lines changed: 6 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -629,7 +629,7 @@ private void flushTerminalNotifications() {
629629
runNotification(notificationForThisThread);
630630
}
631631

632-
// We can not proceed until all of the terminal notifications have executed.
632+
// We can not proceed until all the terminal notifications have executed.
633633
notificationProcessor.doAllWork();
634634
}
635635

@@ -836,6 +836,10 @@ private void computeStatsAndLogCycle(final long cycleTimeNanos) {
836836
}
837837
}
838838

839+
void reportLockWaitNanos(final long lockWaitNanos) {
840+
currentCycleLockWaitTotalNanos += lockWaitNanos;
841+
}
842+
839843
/**
840844
* Is the provided cycle time on budget?
841845
*
@@ -907,7 +911,7 @@ void refreshAllTables() {
907911
private void doRefresh(@NotNull final Runnable refreshFunction) {
908912
final long lockStartTimeNanos = System.nanoTime();
909913
exclusiveLock().doLocked(() -> {
910-
currentCycleLockWaitTotalNanos += System.nanoTime() - lockStartTimeNanos;
914+
reportLockWaitNanos(System.nanoTime() - lockStartTimeNanos);
911915
if (!running) {
912916
return;
913917
}

engine/table/src/main/java/io/deephaven/engine/updategraph/impl/EventDrivenUpdateGraph.java

Lines changed: 15 additions & 7 deletions
Original file line numberDiff line numberDiff line change
@@ -53,14 +53,22 @@ public int parallelismFactor() {
5353
*/
5454
@Override
5555
public void requestRefresh() {
56-
maybeStart();
57-
// do the work to refresh everything, on this thread
58-
isUpdateThread.set(true);
59-
try (final SafeCloseable ignored = ExecutionContext.newBuilder().setUpdateGraph(this).build().open()) {
60-
refreshAllTables();
61-
} finally {
62-
isUpdateThread.remove();
56+
if (isUpdateThread.get()) {
57+
throw new IllegalStateException("Cannot request a refresh from an update thread");
6358
}
59+
maybeStart();
60+
// Do the work to refresh everything, driven by this thread. Note that we acquire the lock "early" in order to
61+
// avoid any inconsistencies w.r.t. assumptions about clock, lock, and update-thread state.
62+
final long lockStartTimeNanos = System.nanoTime();
63+
exclusiveLock().doLocked(() -> {
64+
reportLockWaitNanos(System.nanoTime() - lockStartTimeNanos);
65+
isUpdateThread.set(true);
66+
try (final SafeCloseable ignored = ExecutionContext.newBuilder().setUpdateGraph(this).build().open()) {
67+
refreshAllTables();
68+
} finally {
69+
isUpdateThread.remove();
70+
}
71+
});
6472
final long nowNanos = System.nanoTime();
6573
synchronized (this) {
6674
maybeFlushUpdatePerformance(nowNanos, nowNanos);

engine/table/src/test/java/io/deephaven/engine/updategraph/impl/TestEventDrivenUpdateGraph.java

Lines changed: 56 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -17,15 +17,20 @@
1717
import io.deephaven.engine.table.impl.perf.UpdatePerformanceTracker;
1818
import io.deephaven.engine.table.impl.sources.LongSingleValueSource;
1919
import io.deephaven.engine.testutil.TstUtils;
20+
import io.deephaven.engine.updategraph.TerminalNotification;
2021
import io.deephaven.engine.updategraph.UpdateGraph;
2122
import io.deephaven.engine.util.TableTools;
2223
import io.deephaven.util.SafeCloseable;
2324
import io.deephaven.util.annotations.ReflexiveUse;
2425
import junit.framework.TestCase;
26+
import org.apache.commons.lang3.mutable.MutableInt;
2527
import org.junit.*;
2628

2729
import java.nio.file.Path;
30+
import java.util.ArrayList;
2831
import java.util.Collections;
32+
import java.util.List;
33+
import java.util.concurrent.*;
2934

3035
import static io.deephaven.engine.context.TestExecutionContext.OPERATION_INITIALIZATION;
3136
import static io.deephaven.engine.util.TableTools.*;
@@ -165,6 +170,57 @@ public void testSimpleModify() {
165170
}
166171
}
167172

173+
@Test
174+
public void testRefreshRace() throws ExecutionException, InterruptedException, TimeoutException {
175+
final EventDrivenUpdateGraph eventDrivenUpdateGraph = EventDrivenUpdateGraph.newBuilder("TestEDUG").build();
176+
final List<Runnable> retainedReferences = new ArrayList<>();
177+
178+
final MutableInt sourceRefreshCount = new MutableInt(0);
179+
final Runnable sleepingSource = () -> {
180+
try {
181+
Thread.sleep(100);
182+
sourceRefreshCount.increment();
183+
} catch (InterruptedException e) {
184+
Assert.fail("Interrupted while sleeping");
185+
}
186+
};
187+
retainedReferences.add(sleepingSource);
188+
eventDrivenUpdateGraph.addSource(sleepingSource);
189+
190+
final int numConcurrentRefreshes = 10;
191+
final Future<?>[] refreshFutures = new Future[numConcurrentRefreshes];
192+
final ExecutorService executor = Executors.newFixedThreadPool(numConcurrentRefreshes);
193+
try {
194+
for (int cri = 0; cri < numConcurrentRefreshes; ++cri) {
195+
refreshFutures[cri] = executor.submit(eventDrivenUpdateGraph::requestRefresh);
196+
Thread.sleep(10);
197+
}
198+
for (final Future<?> refreshFuture : refreshFutures) {
199+
refreshFuture.get(10, TimeUnit.SECONDS);
200+
}
201+
} finally {
202+
executor.shutdown();
203+
Assert.assertTrue(executor.awaitTermination(1, TimeUnit.SECONDS));
204+
}
205+
206+
Assert.assertEquals(numConcurrentRefreshes, sourceRefreshCount.intValue());
207+
Assert.assertEquals(sleepingSource, retainedReferences.get(0));
208+
}
209+
210+
@Test
211+
public void testIllegalRefresh() {
212+
final EventDrivenUpdateGraph eventDrivenUpdateGraph = EventDrivenUpdateGraph.newBuilder("TestEDUG").build();
213+
214+
eventDrivenUpdateGraph.addNotification(new TerminalNotification() {
215+
@Override
216+
public void run() {
217+
Assert.assertThrows(IllegalStateException.class, eventDrivenUpdateGraph::requestRefresh);
218+
}
219+
});
220+
221+
eventDrivenUpdateGraph.requestRefresh();
222+
}
223+
168224
@Test
169225
public void testUpdatePerformanceTracker() {
170226
final Table upt = UpdatePerformanceTracker.getQueryTable();

0 commit comments

Comments
 (0)