Skip to content
New issue

Have a question about this project? Sign up for a free GitHub account to open an issue and contact its maintainers and the community.

By clicking “Sign up for GitHub”, you agree to our terms of service and privacy statement. We’ll occasionally send you account related emails.

Already on GitHub? Sign in to your account

feat: Simplify Barrage Viewport Table Updates #6347

Merged
merged 38 commits into from
Nov 18, 2024
Merged
Show file tree
Hide file tree
Changes from 1 commit
Commits
Show all changes
38 commits
Select commit Hold shift + click to select a range
1852b0a
Restore initial commit from grpc-java, plus a few local changes
niloc132 Oct 21, 2024
68b925a
Guard writing payload as hex if FINEST is enabled
niloc132 Oct 21, 2024
e88c47e
Apply upstream "Fix AsyncServletOutputStreamWriterConcurrencyTest
niloc132 Oct 21, 2024
f9a19fc
Apply upstream "Avoid flushing headers when the server returns a single
niloc132 Oct 21, 2024
4733524
Apply upstream "servlet: introduce ServletServerBuilder.buildServlet()"
niloc132 Oct 21, 2024
06e63ec
Bump grpc vers, add inprocess dep for tests
niloc132 Oct 21, 2024
09ade64
Merge branch 'main' into grpc-history-replay
niloc132 Oct 28, 2024
c8af47c
Apply https://github.com/deephaven/deephaven-core/pull/6301
niloc132 Oct 28, 2024
57c8008
Bump to 1.65.1 to better match arrow 18
niloc132 Nov 1, 2024
cbf8ab2
Merge remote-tracking branch 'colin/grpc-history-replay' into vp_simp…
nbauernfeind Nov 6, 2024
85f604f
Version Upgrades; MavenLocal
nbauernfeind Nov 6, 2024
70a0207
Implement Simplified Viewport Table Updates in BMP/BT
nbauernfeind Nov 8, 2024
0089d62
Ryan's Synchronous Review
nbauernfeind Nov 9, 2024
485746d
Merge remote-tracking branch 'upstream/main' into vp_simplification
nbauernfeind Nov 9, 2024
ad8de73
Remove SNAPSHOT version and mavenLocal references
nbauernfeind Nov 11, 2024
02ce2ad
Fixes removed/added rows in most VP cases
nbauernfeind Nov 12, 2024
da23e2b
Bug fixes around viewport snapshot rowsRemoved and rowsAdded
nbauernfeind Nov 12, 2024
299f56e
Bugfix for correct growing VP logic
nbauernfeind Nov 12, 2024
9d6f389
remaining java side fixes
nbauernfeind Nov 13, 2024
fd5aced
Ryan's feedback on javaserver/client impls
nbauernfeind Nov 14, 2024
53b1eed
Inline Feedback from VC w/Ryan
nbauernfeind Nov 14, 2024
6e7fe94
Do not propagate modifies for any repainted rows
nbauernfeind Nov 14, 2024
d568eb7
Minor cleanup from personal review
nbauernfeind Nov 14, 2024
6653ca6
Ryan's feedback latest round.
nbauernfeind Nov 14, 2024
44cdf93
jsAPI mostly complete; looking for tree table issue
nbauernfeind Nov 15, 2024
d549d79
Fixes for jsapi and HierarchicalTable
nbauernfeind Nov 15, 2024
b4d5b69
Lazily compute rowset encoding
nbauernfeind Nov 15, 2024
6c12314
Fixup jsapi tests
nbauernfeind Nov 15, 2024
f9be6e5
Quick round feedback
nbauernfeind Nov 15, 2024
4252622
spotless
nbauernfeind Nov 15, 2024
2767def
Double checked locking fixes
nbauernfeind Nov 15, 2024
78c4cb7
Ryan's final review
nbauernfeind Nov 15, 2024
ea6f898
Clarify strategy on who owns RowSets passed into getSubView
nbauernfeind Nov 15, 2024
3eeb628
npe fix
nbauernfeind Nov 15, 2024
84a6100
Bugfix if HT is empty or viewport past end of table
nbauernfeind Nov 16, 2024
476ae65
Colin's feedback
nbauernfeind Nov 16, 2024
738cb11
Limit jsapi data change event to prev and curr table sizes
nbauernfeind Nov 16, 2024
44fadff
Colin's Final Feedback
nbauernfeind Nov 18, 2024
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
Original file line number Diff line number Diff line change
Expand Up @@ -48,11 +48,14 @@ public static class AddColumnData {
/** The size of the table after this update. -1 if unknown. */
public long tableSize = -1;

public boolean isSnapshot;
/** The RowSet the server is now respecting for this client; only set when parsing on the client. */
public RowSet snapshotRowSet;
/** Whether the server-respecting viewport is a tail; only set when parsing on the client. */
public boolean snapshotRowSetIsReversed;
/** The BitSet of columns the server is now respecting for this client; only set when parsing on the client. */
public BitSet snapshotColumns;

public boolean isSnapshot;
public RowSet rowsAdded;
public RowSet rowsIncluded;
public RowSet rowsRemoved;
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -242,7 +242,7 @@ private final class SubView implements RecordBatchMessageView {
private final boolean isFullSubscription;
private final RowSet viewport;
private final boolean reverseViewport;
private final RowSet keyspaceViewport;
private final boolean hasViewport;
private final BitSet subscribedColumns;

private final long numClientIncludedRows;
Expand All @@ -266,7 +266,7 @@ public SubView(final BarrageSubscriptionOptions options,
this.isFullSubscription = isFullSubscription;
this.viewport = viewport;
this.reverseViewport = reverseViewport;
this.keyspaceViewport = keyspaceViewport;
this.hasViewport = keyspaceViewport != null;
this.subscribedColumns = subscribedColumns;

// precompute the included rows / offsets and viewport removed rows
Expand Down Expand Up @@ -491,7 +491,7 @@ private ByteBuffer getSubscriptionMetadata() throws IOException {
TIntArrayList modOffsets = new TIntArrayList(modColumnData.length);
for (int ii = 0; ii < modColumnData.length; ++ii) {
final int myModRowOffset;
if (keyspaceViewport != null) {
if (hasViewport) {
try (final RowSetGenerator modRowsGen = new RowSetGenerator(clientModdedRows[ii])) {
myModRowOffset = modRowsGen.addToFlatBuffer(metadata);
}
Expand Down Expand Up @@ -1152,7 +1152,7 @@ protected void ensureComputed() throws IOException {
}

try (final ExposedByteArrayOutputStream baos = new ExposedByteArrayOutputStream();
final LittleEndianDataOutputStream oos = new LittleEndianDataOutputStream(baos)) {
final LittleEndianDataOutputStream oos = new LittleEndianDataOutputStream(baos)) {
ExternalizableRowSetUtils.writeExternalCompressedDeltas(oos, original);
oos.flush();
len = baos.size();
Expand Down Expand Up @@ -1181,8 +1181,8 @@ protected int addToFlatBuffer(final RowSet viewport, final FlatBufferBuilder bui
final RowSet viewOfOriginal = original.intersect(viewport)) {
ExternalizableRowSetUtils.writeExternalCompressedDeltas(oos, viewOfOriginal);
oos.flush();
nraw = baos.peekBuffer();
nlen = baos.size();
nraw = baos.peekBuffer();
}

return builder.createByteVector(nraw, 0, nlen);
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -1379,6 +1379,7 @@ private void updateSubscriptionsSnapshotAndPropagate() {
if (!snapshot.rowsAdded.isEmpty()) {
snapshot.rowsAdded.close();
snapshot.rowsAdded = RowSetFactory.empty();
snapshot.tableSize = 0;
}
}
long elapsed = System.nanoTime() - start;
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -323,17 +323,14 @@ private static void buildAndSendSnapshot(
columns, rows, destinations);
snapshotNanosConsumer.accept(System.nanoTime() - snapshotStartNanos);

// note that keyspace is identical to position space for HierarchicalTableView snapshots
final RowSet snapshotRows = RowSetFactory.flat(expandedSize);
final RowSet keyspaceViewportRows = rows.intersect(snapshotRows);

// 4. Make and populate a BarrageMessage
final BarrageMessage barrageMessage = new BarrageMessage();
// We don't populate firstSeq, or lastSeq debugging information; they are not relevant to this use case.
// We don't populate firstSeq or lastSeq debugging information; they are not relevant to this use case.

barrageMessage.isSnapshot = true;
barrageMessage.rowsAdded = snapshotRows;
barrageMessage.rowsIncluded = keyspaceViewportRows;
barrageMessage.rowsAdded = RowSetFactory.flat(expandedSize);
barrageMessage.rowsIncluded = RowSetFactory.fromRange(
rows.firstRowKey(), Math.min(expandedSize - 1, rows.lastRowKey()));
barrageMessage.rowsRemoved = RowSetFactory.empty();
barrageMessage.shifted = RowSetShiftData.EMPTY;
barrageMessage.tableSize = expandedSize;
Expand Down Expand Up @@ -365,11 +362,10 @@ private static void buildAndSendSnapshot(
final boolean isFullSubscription = false;
GrpcUtil.safelyOnNext(listener, streamGenerator.getSubView(
subscriptionOptions, initialSnapshot, isFullSubscription, rows, false,
prevKeyspaceViewportRows, keyspaceViewportRows, columns));
}
prevKeyspaceViewportRows, barrageMessage.rowsIncluded, columns));

prevKeyspaceViewportRows.clear();
prevKeyspaceViewportRows.insert(keyspaceViewportRows);
prevKeyspaceViewportRows.resetTo(barrageMessage.rowsIncluded);
Copy link
Member

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

I think getSubView needs to document that (1) it does not own the row sets passed in, and (2) when they can be mutated.
The keyspace viewports can be mutated as soon as the call returns.
The position space viewport must not be mutated until the message is sent.

}
}

public void setViewport(
Expand Down
Loading