diff --git a/engine/api/src/main/java/io/deephaven/engine/exceptions/IncompatibleTableDefinitionException.java b/engine/api/src/main/java/io/deephaven/engine/exceptions/IncompatibleTableDefinitionException.java
new file mode 100644
index 00000000000..479154bfe5d
--- /dev/null
+++ b/engine/api/src/main/java/io/deephaven/engine/exceptions/IncompatibleTableDefinitionException.java
@@ -0,0 +1,26 @@
+package io.deephaven.engine.exceptions;
+
+import io.deephaven.UncheckedDeephavenException;
+
+/**
+ * Runtime exception representing an incompatibility between table definitions.
+ */
+@SuppressWarnings({"WeakerAccess", "unused"})
+public class IncompatibleTableDefinitionException extends UncheckedDeephavenException {
+
+ public IncompatibleTableDefinitionException() {
+ super();
+ }
+
+ public IncompatibleTableDefinitionException(String message) {
+ super(message);
+ }
+
+ public IncompatibleTableDefinitionException(String message, Throwable cause) {
+ super(message, cause);
+ }
+
+ public IncompatibleTableDefinitionException(Throwable cause) {
+ super(cause);
+ }
+}
diff --git a/engine/api/src/main/java/io/deephaven/engine/table/ColumnSource.java b/engine/api/src/main/java/io/deephaven/engine/table/ColumnSource.java
index 9299fcfca7b..f3486672106 100644
--- a/engine/api/src/main/java/io/deephaven/engine/table/ColumnSource.java
+++ b/engine/api/src/main/java/io/deephaven/engine/table/ColumnSource.java
@@ -50,7 +50,7 @@ WritableRowSet match(
* values until this method is called. This is an option, not an obligation: some simple ColumnSource
* implementations (like TSingleValueSource for various T) always track previous values; other implementations (like
* PrevColumnSource) never do; some (like TArrayColumnSource) only start tracking once this method is called.
- *
+ *
* An immutable column source can not have distinct prev values; therefore it is implemented as a no-op.
*/
default void startTrackingPrevValues() {
@@ -202,7 +202,7 @@ default ColumnSource cast(Class extends TYPE> clazz, @Nullable Cl
/**
* Most column sources will return the same value for a given row without respect to the order that the rows are
* read. Those columns sources are considered "stateless" and should return true.
- *
+ *
* Some column sources, however may be dependent on evaluation order. For example, a formula that updates a Map must
* be evaluated from the first row to the last row. A column source that has the potential to depend on the order of
* evaluation must return false.
diff --git a/engine/api/src/main/java/io/deephaven/engine/table/TableDefinition.java b/engine/api/src/main/java/io/deephaven/engine/table/TableDefinition.java
index 45fe5a33a37..a2903fc4f2f 100644
--- a/engine/api/src/main/java/io/deephaven/engine/table/TableDefinition.java
+++ b/engine/api/src/main/java/io/deephaven/engine/table/TableDefinition.java
@@ -3,13 +3,13 @@
*/
package io.deephaven.engine.table;
-import io.deephaven.UncheckedDeephavenException;
import io.deephaven.api.ColumnName;
import io.deephaven.base.cache.OpenAddressedCanonicalizationCache;
import io.deephaven.base.log.LogOutput;
import io.deephaven.base.log.LogOutputAppendable;
import io.deephaven.base.verify.Assert;
import io.deephaven.engine.table.impl.NoSuchColumnException;
+import io.deephaven.engine.exceptions.IncompatibleTableDefinitionException;
import io.deephaven.io.log.impl.LogOutputStringImpl;
import io.deephaven.qst.column.header.ColumnHeader;
import org.jetbrains.annotations.NotNull;
@@ -654,29 +654,4 @@ private List> getWritableColumns(final boolean partitioningT
}
return columns;
}
-
- /**
- * Runtime exception representing an incompatibility between table definitions.
- */
- @SuppressWarnings({"WeakerAccess", "unused"})
- public static class IncompatibleTableDefinitionException extends UncheckedDeephavenException {
-
- private static final long serialVersionUID = 7668080323885707687L;
-
- public IncompatibleTableDefinitionException() {
- super();
- }
-
- public IncompatibleTableDefinitionException(String message) {
- super(message);
- }
-
- public IncompatibleTableDefinitionException(String message, Throwable cause) {
- super(message, cause);
- }
-
- public IncompatibleTableDefinitionException(Throwable cause) {
- super(cause);
- }
- }
}
diff --git a/engine/base/src/main/java/io/deephaven/engine/table/impl/DefaultChunkSource.java b/engine/base/src/main/java/io/deephaven/engine/table/impl/DefaultChunkSource.java
index fbfdff00865..ef46a04fcb7 100644
--- a/engine/base/src/main/java/io/deephaven/engine/table/impl/DefaultChunkSource.java
+++ b/engine/base/src/main/java/io/deephaven/engine/table/impl/DefaultChunkSource.java
@@ -72,7 +72,7 @@ default Chunk getPrevChunkByFilling(@NotNull final GetContext context,
default ChunkSource getPrevSource() {
final ChunkSource.WithPrev chunkSource = this;
- return new ChunkSource() {
+ return new ChunkSource<>() {
@Override
public ChunkType getChunkType() {
return chunkSource.getChunkType();
diff --git a/engine/table/build.gradle b/engine/table/build.gradle
index 261678a3c06..3732a1dad07 100644
--- a/engine/table/build.gradle
+++ b/engine/table/build.gradle
@@ -37,6 +37,7 @@ dependencies {
implementation project(':plugin')
implementation depCommonsLang3
+ implementation 'org.testng:testng:7.1.0'
Classpaths.inheritCommonsText(project, 'implementation')
Classpaths.inheritGroovy(project, 'groovy', 'implementation')
diff --git a/engine/table/src/main/java/io/deephaven/engine/table/impl/AbstractColumnSource.java b/engine/table/src/main/java/io/deephaven/engine/table/impl/AbstractColumnSource.java
index 889ab074f06..e057aca5c16 100644
--- a/engine/table/src/main/java/io/deephaven/engine/table/impl/AbstractColumnSource.java
+++ b/engine/table/src/main/java/io/deephaven/engine/table/impl/AbstractColumnSource.java
@@ -140,6 +140,7 @@ public WritableRowSet match(
final boolean caseInsensitive,
@NotNull final RowSet mapper,
final Object... keys) {
+ // TODO (deephaven-core#3851): port this to new grouping API
final Map groupToRange = (isImmutable() || !usePrev) ? getGroupToRange(mapper) : null;
if (groupToRange != null) {
RowSetBuilderRandom allInMatchingGroups = RowSetFactory.builderRandom();
diff --git a/engine/table/src/main/java/io/deephaven/engine/table/impl/SourcePartitionedTable.java b/engine/table/src/main/java/io/deephaven/engine/table/impl/SourcePartitionedTable.java
index 4abda3d3786..db7a72cb8e4 100644
--- a/engine/table/src/main/java/io/deephaven/engine/table/impl/SourcePartitionedTable.java
+++ b/engine/table/src/main/java/io/deephaven/engine/table/impl/SourcePartitionedTable.java
@@ -203,7 +203,7 @@ private Table makeConstituentTable(@NotNull final TableLocation tableLocation) {
final PartitionAwareSourceTable constituent = new PartitionAwareSourceTable(
constituentDefinition,
"SingleLocationSourceTable-" + tableLocation,
- RegionedTableComponentFactoryImpl.INSTANCE,
+ RegionedTableComponentFactoryImpl.make(),
new SingleTableLocationProvider(tableLocation),
refreshSizes ? refreshCombiner : null);
diff --git a/engine/table/src/main/java/io/deephaven/engine/table/impl/chunkfilter/ChunkFilter.java b/engine/table/src/main/java/io/deephaven/engine/table/impl/chunkfilter/ChunkFilter.java
index 6fb73115598..25996bab3d2 100644
--- a/engine/table/src/main/java/io/deephaven/engine/table/impl/chunkfilter/ChunkFilter.java
+++ b/engine/table/src/main/java/io/deephaven/engine/table/impl/chunkfilter/ChunkFilter.java
@@ -7,10 +7,12 @@
import io.deephaven.engine.exceptions.CancellationException;
import io.deephaven.engine.rowset.*;
import io.deephaven.engine.rowset.RowSetFactory;
+import io.deephaven.engine.table.ChunkSource;
import io.deephaven.engine.table.ColumnSource;
import io.deephaven.chunk.*;
import io.deephaven.engine.rowset.chunkattributes.OrderedRowKeys;
import io.deephaven.chunk.attributes.Values;
+import org.jetbrains.annotations.NotNull;
public interface ChunkFilter {
/**
@@ -138,14 +140,17 @@ default void filter(Chunk extends Values> values, LongChunk ke
* filter.
*
* @param selection the RowSet to filter
- * @param columnSource the column source to filter
+ * @param chunkSource the chunk source to filter
* @param usePrev should we use previous values from the column source?
* @param chunkFilter the chunk filter to apply
*
* @return A new WritableRowSet representing the filtered values, owned by the caller
*/
- static WritableRowSet applyChunkFilter(RowSet selection, ColumnSource> columnSource, boolean usePrev,
- ChunkFilter chunkFilter) {
+ static WritableRowSet applyChunkFilter(
+ @NotNull final RowSet selection,
+ @NotNull final ChunkSource extends Values> chunkSource,
+ final boolean usePrev,
+ @NotNull final ChunkFilter chunkFilter) {
final RowSetBuilderSequential builder = RowSetFactory.builderSequential();
final int contextSize = (int) Math.min(FILTER_CHUNK_SIZE, selection.size());
@@ -153,7 +158,7 @@ static WritableRowSet applyChunkFilter(RowSet selection, ColumnSource> columnS
long filteredChunks = 0;
long lastInterruptCheck = System.currentTimeMillis();
- try (final ColumnSource.GetContext getContext = columnSource.makeGetContext(contextSize);
+ try (final ColumnSource.GetContext getContext = chunkSource.makeGetContext(contextSize);
final WritableLongChunk longChunk = WritableLongChunk.makeWritableChunk(contextSize);
final RowSequence.Iterator rsIt = selection.getRowSequenceIterator()) {
while (rsIt.hasMore()) {
@@ -176,9 +181,11 @@ static WritableRowSet applyChunkFilter(RowSet selection, ColumnSource> columnS
final Chunk extends Values> dataChunk;
if (usePrev) {
- dataChunk = columnSource.getPrevChunk(getContext, okChunk);
+ // noinspection unchecked
+ dataChunk = ((ChunkSource.WithPrev extends Values>) chunkSource)
+ .getPrevChunk(getContext, okChunk);
} else {
- dataChunk = columnSource.getChunk(getContext, okChunk);
+ dataChunk = chunkSource.getChunk(getContext, okChunk);
}
chunkFilter.filter(dataChunk, keyChunk, longChunk);
diff --git a/engine/table/src/main/java/io/deephaven/engine/table/impl/locations/ColumnLocation.java b/engine/table/src/main/java/io/deephaven/engine/table/impl/locations/ColumnLocation.java
index fe6073508bf..6bf38ede4f6 100644
--- a/engine/table/src/main/java/io/deephaven/engine/table/impl/locations/ColumnLocation.java
+++ b/engine/table/src/main/java/io/deephaven/engine/table/impl/locations/ColumnLocation.java
@@ -41,6 +41,16 @@ public interface ColumnLocation extends StringUtils.StringKeyedObject, NamedImpl
*/
boolean exists();
+ /**
+ * Get this column location cast to the specified type
+ *
+ * @return {@code this}, with the appropriate cast applied
+ */
+ default CL cast() {
+ // noinspection unchecked
+ return (CL) this;
+ }
+
/**
*
* Get the metadata object stored with this column, or null if no such data exists.
diff --git a/engine/table/src/main/java/io/deephaven/engine/table/impl/locations/KeyRangeGroupingProvider.java b/engine/table/src/main/java/io/deephaven/engine/table/impl/locations/KeyRangeGroupingProvider.java
index fbf2d98f2c4..a8695e8d286 100644
--- a/engine/table/src/main/java/io/deephaven/engine/table/impl/locations/KeyRangeGroupingProvider.java
+++ b/engine/table/src/main/java/io/deephaven/engine/table/impl/locations/KeyRangeGroupingProvider.java
@@ -4,7 +4,6 @@
package io.deephaven.engine.table.impl.locations;
import io.deephaven.engine.rowset.RowSet;
-import io.deephaven.engine.rowset.TrackingWritableRowSet;
import org.jetbrains.annotations.NotNull;
/**
diff --git a/engine/table/src/main/java/io/deephaven/engine/table/impl/locations/TableLocation.java b/engine/table/src/main/java/io/deephaven/engine/table/impl/locations/TableLocation.java
index a4fb0085240..b086f0d6d9b 100644
--- a/engine/table/src/main/java/io/deephaven/engine/table/impl/locations/TableLocation.java
+++ b/engine/table/src/main/java/io/deephaven/engine/table/impl/locations/TableLocation.java
@@ -3,12 +3,17 @@
*/
package io.deephaven.engine.table.impl.locations;
+import io.deephaven.api.SortColumn;
import io.deephaven.base.log.LogOutput;
import io.deephaven.base.log.LogOutputAppendable;
+import io.deephaven.engine.table.Table;
import io.deephaven.io.log.impl.LogOutputStringImpl;
import io.deephaven.util.annotations.FinalDefault;
import io.deephaven.util.type.NamedImplementation;
import org.jetbrains.annotations.NotNull;
+import org.jetbrains.annotations.Nullable;
+
+import java.util.List;
/**
* Building block for Deephaven "source" tables, with helper methods for discovering locations and their sizes. A
@@ -36,6 +41,12 @@ interface Listener extends BasicTableDataListener {
@NotNull
ImmutableTableKey getTableKey();
+ // TODO: NATE NOCOMMIT IF UNUSED ELSE MAKE CAST
+ default T as(Class otherType) {
+ // noinspection unchecked
+ return (T) this;
+ }
+
/**
* @return An {@link ImmutableTableLocationKey} instance for this location
*/
@@ -77,6 +88,31 @@ interface Listener extends BasicTableDataListener {
*/
void refresh();
+ /**
+ * Get an ordered list of columns this location is sorted by.
+ *
+ * @return a non-null ordered list of {@link SortColumn}s
+ */
+ @NotNull
+ List getSortedColumns();
+
+ /**
+ * Check if this location has a data index for the specified columns.
+ *
+ * @param columns the set of columns to check for.
+ * @return true if the table has a Data Index for the specified columns
+ */
+ boolean hasDataIndexFor(@NotNull String... columns);
+
+ /**
+ * Get the data index table for the specified set of columns. Note that the order of columns does not matter here.
+ *
+ * @param columns the key columns for the index
+ * @return the index table or null if one does not exist.
+ */
+ @Nullable
+ Table getDataIndex(@NotNull String... columns);
+
/**
* @param name The column name
* @return The ColumnLocation for the defined column under this table location
diff --git a/engine/table/src/main/java/io/deephaven/engine/table/impl/locations/impl/AbstractTableLocation.java b/engine/table/src/main/java/io/deephaven/engine/table/impl/locations/impl/AbstractTableLocation.java
index 85ddd813832..04e1c226904 100644
--- a/engine/table/src/main/java/io/deephaven/engine/table/impl/locations/impl/AbstractTableLocation.java
+++ b/engine/table/src/main/java/io/deephaven/engine/table/impl/locations/impl/AbstractTableLocation.java
@@ -4,11 +4,20 @@
package io.deephaven.engine.table.impl.locations.impl;
import io.deephaven.base.verify.Require;
+import io.deephaven.engine.table.Table;
import io.deephaven.engine.util.string.StringUtils;
import io.deephaven.engine.table.impl.locations.*;
import io.deephaven.engine.rowset.RowSet;
import io.deephaven.hash.KeyedObjectHashMap;
import org.jetbrains.annotations.NotNull;
+import org.jetbrains.annotations.Nullable;
+
+import java.lang.ref.SoftReference;
+import java.util.Arrays;
+import java.util.HashMap;
+import java.util.List;
+import java.util.Map;
+import java.util.concurrent.atomic.AtomicLong;
/**
* Partial TableLocation implementation for use by TableDataService implementations.
@@ -16,6 +25,7 @@
public abstract class AbstractTableLocation
extends SubscriptionAggregator
implements TableLocation {
+ protected static final SoftReference NO_GROUPING_SENTINEL = new SoftReference<>(null);
private final ImmutableTableKey tableKey;
private final ImmutableTableLocationKey tableLocationKey;
@@ -24,6 +34,11 @@ public abstract class AbstractTableLocation
private final KeyedObjectHashMap columnLocations =
new KeyedObjectHashMap<>(StringUtils.charSequenceKey());
+ /**
+ * A map of grouping (or data index) columns to the materialized
+ */
+ protected volatile Map, SoftReference> cachedGroupings;
+
/**
* @param tableKey Table key for the table this location belongs to
* @param tableLocationKey Table location key that identifies this location
@@ -138,4 +153,63 @@ public final ColumnLocation getColumnLocation(@NotNull final CharSequence name)
protected final void clearColumnLocations() {
columnLocations.clear();
}
+
+ @Nullable
+ @Override
+ public final Table getDataIndex(@NotNull final String... columns) {
+ final List colNames = Arrays.asList(columns);
+ Table grouping = null;
+ if (cachedGroupings != null) {
+ final SoftReference cachedGrouping = cachedGroupings.get(colNames);
+ if (cachedGrouping == NO_GROUPING_SENTINEL) {
+ return null;
+ }
+
+ if (cachedGrouping != null) {
+ grouping = cachedGrouping.get();
+ if (grouping != null) {
+ // System.out.println("HAD CACHE");
+ return grouping;
+ }
+ }
+ }
+
+ synchronized (this) {
+ if (cachedGroupings == null) {
+ cachedGroupings = new HashMap<>();
+ }
+
+ final SoftReference cachedGrouping = cachedGroupings.get(colNames);
+ if (cachedGrouping == NO_GROUPING_SENTINEL) {
+ return null;
+ }
+
+ if (cachedGrouping != null) {
+ grouping = cachedGrouping.get();
+ }
+
+ if (grouping == null) {
+ grouping = getDataIndexImpl(columns);
+
+ if (grouping == null || grouping.isEmpty()) {
+ cachedGroupings.put(colNames, NO_GROUPING_SENTINEL);
+ } else {
+ // System.out.println("NO CACHE");
+ cachedGroupings.put(colNames, new SoftReference<>(grouping));
+ }
+ }
+
+ return grouping;
+ }
+ }
+
+ /**
+ * Load the data index from the location implementation. Implementations of this method should not perform any
+ * result caching.
+ *
+ * @param columns the columns to load an index for.
+ * @return the data index table, or an empty table or null if none existed.
+ */
+ @Nullable
+ protected abstract Table getDataIndexImpl(@NotNull final String... columns);
}
diff --git a/engine/table/src/main/java/io/deephaven/engine/table/impl/locations/impl/NonexistentTableLocation.java b/engine/table/src/main/java/io/deephaven/engine/table/impl/locations/impl/NonexistentTableLocation.java
index 9bca3b3087f..15ca82055ce 100644
--- a/engine/table/src/main/java/io/deephaven/engine/table/impl/locations/impl/NonexistentTableLocation.java
+++ b/engine/table/src/main/java/io/deephaven/engine/table/impl/locations/impl/NonexistentTableLocation.java
@@ -3,12 +3,17 @@
*/
package io.deephaven.engine.table.impl.locations.impl;
+import io.deephaven.api.SortColumn;
+import io.deephaven.engine.table.Table;
import io.deephaven.engine.table.impl.locations.ColumnLocation;
import io.deephaven.engine.table.impl.locations.TableKey;
import io.deephaven.engine.table.impl.locations.TableLocation;
import io.deephaven.engine.table.impl.locations.TableLocationKey;
import io.deephaven.engine.rowset.RowSetFactory;
import org.jetbrains.annotations.NotNull;
+import org.jetbrains.annotations.Nullable;
+
+import java.util.List;
/**
* {@link TableLocation} implementation for locations that are found to not actually exist when accessed.
@@ -36,4 +41,20 @@ public void refresh() {}
protected ColumnLocation makeColumnLocation(@NotNull String name) {
throw new UnsupportedOperationException();
}
+
+ @Override
+ public @NotNull List getSortedColumns() {
+ // TODO NATE NOCOMMIT: what is ideal here?
+ throw new UnsupportedOperationException();
+ }
+
+ @Override
+ public boolean hasDataIndexFor(@NotNull String... columns) {
+ return false;
+ }
+
+ @Override
+ protected @Nullable Table getDataIndexImpl(@NotNull String... columns) {
+ return null;
+ }
}
diff --git a/engine/table/src/main/java/io/deephaven/engine/table/impl/select/MatchFilter.java b/engine/table/src/main/java/io/deephaven/engine/table/impl/select/MatchFilter.java
index 74fb3080b2b..51b470c5744 100644
--- a/engine/table/src/main/java/io/deephaven/engine/table/impl/select/MatchFilter.java
+++ b/engine/table/src/main/java/io/deephaven/engine/table/impl/select/MatchFilter.java
@@ -13,6 +13,8 @@
import io.deephaven.engine.table.impl.preview.DisplayWrapper;
import io.deephaven.engine.context.QueryScope;
import io.deephaven.time.DateTimeUtils;
+import io.deephaven.util.BooleanUtils;
+import io.deephaven.util.QueryConstants;
import io.deephaven.util.type.ArrayTypeUtils;
import io.deephaven.engine.table.ColumnSource;
import io.deephaven.engine.rowset.RowSet;
@@ -206,6 +208,9 @@ public static ColumnTypeConvertor getConvertor(final Class> cls, final String
return new ColumnTypeConvertor() {
@Override
Object convertStringLiteral(String str) {
+ if ("null".equals(str) || "NULL_BYTE".equals(str)) {
+ return QueryConstants.NULL_BYTE;
+ }
return Byte.parseByte(str);
}
};
@@ -214,6 +219,9 @@ Object convertStringLiteral(String str) {
return new ColumnTypeConvertor() {
@Override
Object convertStringLiteral(String str) {
+ if ("null".equals(str) || "NULL_SHORT".equals(str)) {
+ return QueryConstants.NULL_SHORT;
+ }
return Short.parseShort(str);
}
};
@@ -222,6 +230,9 @@ Object convertStringLiteral(String str) {
return new ColumnTypeConvertor() {
@Override
Object convertStringLiteral(String str) {
+ if ("null".equals(str) || "NULL_INT".equals(str)) {
+ return QueryConstants.NULL_INT;
+ }
return Integer.parseInt(str);
}
};
@@ -230,6 +241,9 @@ Object convertStringLiteral(String str) {
return new ColumnTypeConvertor() {
@Override
Object convertStringLiteral(String str) {
+ if ("null".equals(str) || "NULL_LONG".equals(str)) {
+ return QueryConstants.NULL_LONG;
+ }
return Long.parseLong(str);
}
};
@@ -238,6 +252,9 @@ Object convertStringLiteral(String str) {
return new ColumnTypeConvertor() {
@Override
Object convertStringLiteral(String str) {
+ if ("null".equals(str) || "NULL_FLOAT".equals(str)) {
+ return QueryConstants.NULL_FLOAT;
+ }
return Float.parseFloat(str);
}
};
@@ -246,6 +263,9 @@ Object convertStringLiteral(String str) {
return new ColumnTypeConvertor() {
@Override
Object convertStringLiteral(String str) {
+ if ("null".equals(str) || "NULL_DOUBLE".equals(str)) {
+ return QueryConstants.NULL_DOUBLE;
+ }
return Double.parseDouble(str);
}
};
@@ -261,6 +281,9 @@ Object convertStringLiteral(String str) {
if (str.equalsIgnoreCase("false")) {
return Boolean.FALSE;
}
+ if ("null".equals(str) || "NULL_BOOLEAN".equals(str)) {
+ return BooleanUtils.NULL_BOOLEAN_AS_BYTE;
+ }
throw new IllegalArgumentException("String " + str
+ " isn't a valid boolean value (!str.equalsIgnoreCase(\"true\") && !str.equalsIgnoreCase(\"false\"))");
}
@@ -271,6 +294,9 @@ Object convertStringLiteral(String str) {
@Override
Object convertStringLiteral(String str) {
if (str.length() > 1) {
+ if ("null".equals(str) || "NULL_CHAR".equals(str)) {
+ return QueryConstants.NULL_CHAR;
+ }
// TODO: #1517 Allow escaping of chars
if (str.length() == 3 && ((str.charAt(0) == '\'' && str.charAt(2) == '\'')
|| (str.charAt(0) == '"' && str.charAt(2) == '"'))) {
diff --git a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/ReverseLookupColumnSource.java b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/ReverseLookupColumnSource.java
deleted file mode 100644
index 4c6c7668bf6..00000000000
--- a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/ReverseLookupColumnSource.java
+++ /dev/null
@@ -1,44 +0,0 @@
-/**
- * Copyright (c) 2016-2022 Deephaven Data Labs and Patent Pending
- */
-package io.deephaven.engine.table.impl.sources;
-
-import io.deephaven.stringset.LongBitmapStringSet;
-import io.deephaven.engine.table.ColumnSource;
-
-import java.util.function.ToIntFunction;
-
-/**
- * Common interface for column sources that provide a reverse-lookup function (value to int key). Note that int keys are
- * used because this is intended for column sources with a small, contiguous key range starting from 0 and well shorter
- * than Integer.MAX_VALUE.
- */
-public interface ReverseLookupColumnSource extends ColumnSource,
- LongBitmapStringSet.ReversibleLookup {
- /**
- * Get a reverse-lookup function for all non-null values stored in this column source at
- * {@code keys <= highestKeyNeeded}.
- *
- * @param highestKeyNeeded The highest key needed in the result map
- * @return A reverse-lookup function that has all values defined for keys in [0, highestKeyNeeded]
- */
- ToIntFunction getReverseLookup(final int highestKeyNeeded);
-
- /**
- * Get an implementation-defined "extra value" associated with this column source.
- */
- EXTRA_VALUE_TYPE getExtra();
-
- /**
- * Perform a reverse lookup
- *
- * @param highestIndex The highest key needed for the lookup
- * @param value The value we are looking up
- * @return The key, between 0 and highestIndex, for the value. A value outside this range if the value has no
- * mapping in the range.
- */
-
- default int rget(int highestIndex, DATA_TYPE value) {
- return getReverseLookup(highestIndex).applyAsInt(value);
- }
-}
diff --git a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/ColumnRegion.java b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/ColumnRegion.java
index 718b9056bfd..34cfe1b6685 100644
--- a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/ColumnRegion.java
+++ b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/ColumnRegion.java
@@ -3,15 +3,25 @@
*/
package io.deephaven.engine.table.impl.sources.regioned;
+import io.deephaven.engine.rowset.RowSequence;
+import io.deephaven.engine.rowset.RowSet;
+import io.deephaven.engine.rowset.WritableRowSet;
import io.deephaven.engine.table.Releasable;
import io.deephaven.chunk.attributes.Any;
import io.deephaven.chunk.WritableChunk;
import io.deephaven.engine.page.Page;
+import io.deephaven.engine.table.impl.locations.ColumnLocation;
import io.deephaven.util.annotations.FinalDefault;
import org.jetbrains.annotations.NotNull;
+import org.jetbrains.annotations.Nullable;
public interface ColumnRegion extends Page, Releasable {
+ // TODO NATE NOCOMMIT MAYBE THIS SHOULD BE NULLABLE to better support null regions (and make them singletons again)
+ // javadoc why / when nullable
+ @Nullable
+ ColumnLocation getLocation();
+
@Override
@FinalDefault
default long firstRowOffset() {
@@ -23,6 +33,36 @@ default long firstRowOffset() {
*/
void invalidate();
+ /**
+ * Check if this region has matching features.
+ *
+ * @return true if this has matching features
+ */
+ default boolean supportsMatching() {
+ return false;
+ }
+
+ // Should this really return a writable row set/
+ /**
+ * If {@link #supportsMatching()} is {@code true} this method will perform matching in lieu of the standard
+ * {@link RegionedColumnSourceBase#match(boolean, boolean, boolean, RowSet, Object...)}.
+ *
+ * @param invertMatch if the match should be inverted
+ * @param usePrev if the match should use the previous value
+ * @param caseInsensitive if the match is case insensitive
+ * @param rowSequence relevant rows to search
+ * @param sortedKeys the keys to search for in sorted order
+ * @return an index of matched rows
+ */
+ default WritableRowSet match(
+ boolean invertMatch,
+ boolean usePrev,
+ boolean caseInsensitive,
+ @NotNull RowSequence rowSequence,
+ Object... sortedKeys) {
+ throw new UnsupportedOperationException("This region does not support matching");
+ }
+
abstract class Null
extends GenericColumnRegionBase
implements ColumnRegion, WithDefaultsForRepeatingValues {
@@ -39,5 +79,23 @@ public void fillChunkAppend(@NotNull final FillContext context,
destination.fillWithNullValue(offset, length);
destination.setSize(offset + length);
}
+
+ @Override
+ public final ColumnLocation getLocation() {
+ return null;
+ }
+
+ @Override
+ public final boolean supportsMatching() {
+ return true;
+ }
+
+ @Override
+ public abstract WritableRowSet match(
+ boolean invertMatch,
+ boolean usePrev,
+ boolean caseInsensitive,
+ @NotNull RowSequence rowSequence,
+ Object... sortedKeys);
}
}
diff --git a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/ColumnRegionByte.java b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/ColumnRegionByte.java
index a8f00446713..0cbe37b4297 100644
--- a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/ColumnRegionByte.java
+++ b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/ColumnRegionByte.java
@@ -3,9 +3,14 @@
*/
package io.deephaven.engine.table.impl.sources.regioned;
-import io.deephaven.chunk.attributes.Any;
import io.deephaven.chunk.ChunkType;
import io.deephaven.chunk.WritableChunk;
+import io.deephaven.chunk.attributes.Any;
+import io.deephaven.engine.rowset.RowSequence;
+import io.deephaven.engine.rowset.RowSet;
+import io.deephaven.engine.rowset.RowSetFactory;
+import io.deephaven.engine.rowset.WritableRowSet;
+import io.deephaven.engine.table.impl.locations.ColumnLocation;
import io.deephaven.util.QueryConstants;
import io.deephaven.util.annotations.FinalDefault;
import org.jetbrains.annotations.NotNull;
@@ -82,6 +87,23 @@ public byte[] getBytes(final long firstElementIndex, @NotNull final byte[] desti
Arrays.fill(destination, destinationOffset, destinationOffset + length, QueryConstants.NULL_BYTE);
return destination;
}
+
+ @Override
+ public WritableRowSet match(
+ final boolean invertMatch,
+ final boolean usePrev,
+ final boolean caseInsensitive,
+ @NotNull final RowSequence rowSequence,
+ final Object... sortedKeys) {
+ final boolean nullMatched = sortedKeys.length > 0
+ && (sortedKeys[0] == null || sortedKeys[0].equals(QueryConstants.NULL_BYTE_BOXED));
+ if (nullMatched && !invertMatch || !nullMatched && invertMatch) {
+ try (final RowSet rowSet = rowSequence.asRowSet()) {
+ return rowSet.copy();
+ }
+ }
+ return RowSetFactory.empty();
+ }
}
final class Constant
@@ -112,14 +134,52 @@ public byte[] getBytes(final long firstElementIndex, @NotNull final byte[] desti
Arrays.fill(destination, destinationOffset, destinationOffset + length, value);
return destination;
}
+
+ @Override
+ public ColumnLocation getLocation() {
+ return null;
+ }
+
+ @Override
+ public boolean supportsMatching() {
+ return true;
+ }
+
+ @Override
+ public WritableRowSet match(
+ final boolean invertMatch,
+ final boolean usePrev,
+ final boolean caseInsensitive,
+ @NotNull final RowSequence rowSequence,
+ final Object... sortedKeys) {
+ boolean valueMatches = arrayContainsValue(sortedKeys);
+ if (valueMatches && !invertMatch || !valueMatches && invertMatch) {
+ try (final RowSet rowSet = rowSequence.asRowSet()) {
+ return rowSet.copy();
+ }
+ }
+
+ return RowSetFactory.empty();
+ }
+
+ private boolean arrayContainsValue(final Object[] sortedKeys) {
+ if (value == QueryConstants.NULL_BYTE && sortedKeys.length > 0
+ && (sortedKeys[0] == null || sortedKeys[0] == QueryConstants.NULL_BYTE_BOXED)) {
+ return true;
+ }
+ return Arrays.binarySearch(sortedKeys, value) >= 0;
+ }
}
final class StaticPageStore
extends RegionedPageStore.Static>
implements ColumnRegionByte {
- public StaticPageStore(@NotNull final Parameters parameters, @NotNull final ColumnRegionByte[] regions) {
- super(parameters, regions);
+ public StaticPageStore(
+ @NotNull final Parameters parameters,
+ @NotNull final ColumnRegionByte[] regions,
+ @NotNull final ColumnLocation location) {
+ super(parameters, regions, location);
}
@Override
@@ -143,5 +203,15 @@ public byte getByte(@NotNull final FillContext context, final long elementIndex)
public byte[] getBytes(final long firstElementIndex, @NotNull final byte[] destination, final int destinationOffset, final int length) {
return lookupRegion(firstElementIndex).getBytes(firstElementIndex, destination, destinationOffset, length);
}
+
+ @Override
+ public WritableRowSet match(
+ final boolean invertMatch,
+ final boolean usePrev, boolean caseInsensitive,
+ @NotNull final RowSequence rowSequence,
+ final Object... sortedKeys) {
+ // where is the data stored here?
+ throw new UnsupportedOperationException("TODO NATE NOCOMMIT");
+ }
}
}
diff --git a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/ColumnRegionChar.java b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/ColumnRegionChar.java
index f990541652e..70f255923be 100644
--- a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/ColumnRegionChar.java
+++ b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/ColumnRegionChar.java
@@ -3,13 +3,20 @@
*/
package io.deephaven.engine.table.impl.sources.regioned;
-import io.deephaven.chunk.attributes.Any;
import io.deephaven.chunk.ChunkType;
import io.deephaven.chunk.WritableChunk;
+import io.deephaven.chunk.attributes.Any;
+import io.deephaven.engine.rowset.RowSequence;
+import io.deephaven.engine.rowset.RowSet;
+import io.deephaven.engine.rowset.RowSetFactory;
+import io.deephaven.engine.rowset.WritableRowSet;
+import io.deephaven.engine.table.impl.locations.ColumnLocation;
import io.deephaven.util.QueryConstants;
import io.deephaven.util.annotations.FinalDefault;
import org.jetbrains.annotations.NotNull;
+import java.util.Arrays;
+
/**
* Column region interface for regions that support fetching primitive chars.
*/
@@ -58,6 +65,23 @@ private Null(final long pageMask) {
public char getChar(final long elementIndex) {
return QueryConstants.NULL_CHAR;
}
+
+ @Override
+ public WritableRowSet match(
+ final boolean invertMatch,
+ final boolean usePrev,
+ final boolean caseInsensitive,
+ @NotNull final RowSequence rowSequence,
+ final Object... sortedKeys) {
+ final boolean nullMatched = sortedKeys.length > 0
+ && (sortedKeys[0] == null || sortedKeys[0].equals(QueryConstants.NULL_CHAR_BOXED));
+ if (nullMatched && !invertMatch || !nullMatched && invertMatch) {
+ try (final RowSet rowSet = rowSequence.asRowSet()) {
+ return rowSet.copy();
+ }
+ }
+ return RowSetFactory.empty();
+ }
}
final class Constant
@@ -82,14 +106,52 @@ public void fillChunkAppend(@NotNull final FillContext context, @NotNull final W
destination.asWritableCharChunk().fillWithValue(offset, length, value);
destination.setSize(offset + length);
}
+
+ @Override
+ public ColumnLocation getLocation() {
+ return null;
+ }
+
+ @Override
+ public boolean supportsMatching() {
+ return true;
+ }
+
+ @Override
+ public WritableRowSet match(
+ final boolean invertMatch,
+ final boolean usePrev,
+ final boolean caseInsensitive,
+ @NotNull final RowSequence rowSequence,
+ final Object... sortedKeys) {
+ boolean valueMatches = arrayContainsValue(sortedKeys);
+ if (valueMatches && !invertMatch || !valueMatches && invertMatch) {
+ try (final RowSet rowSet = rowSequence.asRowSet()) {
+ return rowSet.copy();
+ }
+ }
+
+ return RowSetFactory.empty();
+ }
+
+ private boolean arrayContainsValue(final Object[] sortedKeys) {
+ if (value == QueryConstants.NULL_CHAR && sortedKeys.length > 0
+ && (sortedKeys[0] == null || sortedKeys[0] == QueryConstants.NULL_CHAR_BOXED)) {
+ return true;
+ }
+ return Arrays.binarySearch(sortedKeys, value) >= 0;
+ }
}
final class StaticPageStore
extends RegionedPageStore.Static>
implements ColumnRegionChar {
- public StaticPageStore(@NotNull final Parameters parameters, @NotNull final ColumnRegionChar[] regions) {
- super(parameters, regions);
+ public StaticPageStore(
+ @NotNull final Parameters parameters,
+ @NotNull final ColumnRegionChar[] regions,
+ @NotNull final ColumnLocation location) {
+ super(parameters, regions, location);
}
@Override
@@ -108,5 +170,15 @@ public char getChar(final long elementIndex) {
public char getChar(@NotNull final FillContext context, final long elementIndex) {
return lookupRegion(elementIndex).getChar(context, elementIndex);
}
+
+ @Override
+ public WritableRowSet match(
+ final boolean invertMatch,
+ final boolean usePrev, boolean caseInsensitive,
+ @NotNull final RowSequence rowSequence,
+ final Object... sortedKeys) {
+ // where is the data stored here?
+ throw new UnsupportedOperationException("TODO NATE NOCOMMIT");
+ }
}
}
diff --git a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/ColumnRegionChunkDictionary.java b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/ColumnRegionChunkDictionary.java
index d31b33db4c1..ab2fc32961b 100644
--- a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/ColumnRegionChunkDictionary.java
+++ b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/ColumnRegionChunkDictionary.java
@@ -4,9 +4,15 @@
package io.deephaven.engine.table.impl.sources.regioned;
import io.deephaven.base.string.cache.StringCache;
+import io.deephaven.chunk.attributes.Any;
+import io.deephaven.chunk.attributes.Values;
+import io.deephaven.engine.rowset.WritableRowSet;
+import io.deephaven.engine.table.ChunkSource;
import io.deephaven.engine.table.impl.DefaultChunkSource;
+import io.deephaven.engine.table.impl.chunkfilter.ChunkFilter;
+import io.deephaven.engine.table.impl.chunkfilter.ChunkMatchFilterFactory;
+import io.deephaven.engine.table.impl.locations.ColumnLocation;
import io.deephaven.engine.util.string.StringUtils;
-import io.deephaven.chunk.attributes.Any;
import io.deephaven.chunk.*;
import io.deephaven.engine.page.Page;
import io.deephaven.engine.rowset.RowSequence;
@@ -28,11 +34,14 @@ public class ColumnRegionChunkDictionary
private final Supplier> dictionaryChunkSupplier;
private final Function conversion;
+ @NotNull
+ private final ColumnLocation location;
public static ColumnRegionObject create(
final long pageMask,
@NotNull final Class dataType,
- @NotNull final Supplier> dictionaryChunkSupplier) {
+ @NotNull final Supplier> dictionaryChunkSupplier,
+ @NotNull final ColumnLocation location) {
if (CharSequence.class.isAssignableFrom(dataType)) {
// noinspection unchecked
final StringCache> stringCache =
@@ -40,18 +49,21 @@ public static ColumnRegionObject
// noinspection unchecked
final Function conversion =
(final String dictValue) -> (DATA_TYPE) stringCache.getCachedString(dictValue);
- return new ColumnRegionChunkDictionary<>(pageMask, dictionaryChunkSupplier, conversion);
+ return new ColumnRegionChunkDictionary<>(pageMask, dictionaryChunkSupplier, conversion, location);
}
return new ColumnRegionChunkDictionary<>(pageMask, dictionaryChunkSupplier,
- Function.identity());
+ Function.identity(), location);
}
- private ColumnRegionChunkDictionary(final long pageMask,
+ private ColumnRegionChunkDictionary(
+ final long pageMask,
@NotNull final Supplier> dictionaryChunkSupplier,
- @NotNull final Function conversion) {
+ @NotNull final Function conversion,
+ @NotNull final ColumnLocation location) {
super(pageMask);
this.dictionaryChunkSupplier = dictionaryChunkSupplier;
this.conversion = conversion;
+ this.location = location;
}
private ObjectChunk getDictionaryChunk() {
@@ -102,4 +114,29 @@ public boolean gatherDictionaryValuesRowSet(
}
return advanceToNextPage(keysToVisit);
}
+
+ @Override
+ public ColumnLocation getLocation() {
+ return location;
+ }
+
+ @Override
+ public boolean supportsMatching() {
+ return true;
+ }
+
+ @Override
+ public WritableRowSet match(
+ final boolean invertMatch,
+ final boolean usePrev,
+ final boolean caseInsensitive,
+ @NotNull final RowSequence rowSequence,
+ final Object... sortedKeys) {
+ // TODO NATE NOCOMMIT - needs getNativeType (maybe?) pushdown instead?
+ try (RowSet selection = rowSequence.asRowSet()) {
+ // noinspection unchecked
+ return ChunkFilter.applyChunkFilter(selection, (ChunkSource extends Values>) this, false,
+ ChunkMatchFilterFactory.getChunkFilter(Object.class, caseInsensitive, invertMatch, sortedKeys));
+ }
+ }
}
diff --git a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/ColumnRegionDouble.java b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/ColumnRegionDouble.java
index a3445cf2a89..0476bb79ade 100644
--- a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/ColumnRegionDouble.java
+++ b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/ColumnRegionDouble.java
@@ -8,13 +8,20 @@
*/
package io.deephaven.engine.table.impl.sources.regioned;
-import io.deephaven.chunk.attributes.Any;
import io.deephaven.chunk.ChunkType;
import io.deephaven.chunk.WritableChunk;
+import io.deephaven.chunk.attributes.Any;
+import io.deephaven.engine.rowset.RowSequence;
+import io.deephaven.engine.rowset.RowSet;
+import io.deephaven.engine.rowset.RowSetFactory;
+import io.deephaven.engine.rowset.WritableRowSet;
+import io.deephaven.engine.table.impl.locations.ColumnLocation;
import io.deephaven.util.QueryConstants;
import io.deephaven.util.annotations.FinalDefault;
import org.jetbrains.annotations.NotNull;
+import java.util.Arrays;
+
/**
* Column region interface for regions that support fetching primitive doubles.
*/
@@ -63,6 +70,23 @@ private Null(final long pageMask) {
public double getDouble(final long elementIndex) {
return QueryConstants.NULL_DOUBLE;
}
+
+ @Override
+ public WritableRowSet match(
+ final boolean invertMatch,
+ final boolean usePrev,
+ final boolean caseInsensitive,
+ @NotNull final RowSequence rowSequence,
+ final Object... sortedKeys) {
+ final boolean nullMatched = sortedKeys.length > 0
+ && (sortedKeys[0] == null || sortedKeys[0].equals(QueryConstants.NULL_DOUBLE_BOXED));
+ if (nullMatched && !invertMatch || !nullMatched && invertMatch) {
+ try (final RowSet rowSet = rowSequence.asRowSet()) {
+ return rowSet.copy();
+ }
+ }
+ return RowSetFactory.empty();
+ }
}
final class Constant
@@ -87,14 +111,52 @@ public void fillChunkAppend(@NotNull final FillContext context, @NotNull final W
destination.asWritableDoubleChunk().fillWithValue(offset, length, value);
destination.setSize(offset + length);
}
+
+ @Override
+ public ColumnLocation getLocation() {
+ return null;
+ }
+
+ @Override
+ public boolean supportsMatching() {
+ return true;
+ }
+
+ @Override
+ public WritableRowSet match(
+ final boolean invertMatch,
+ final boolean usePrev,
+ final boolean caseInsensitive,
+ @NotNull final RowSequence rowSequence,
+ final Object... sortedKeys) {
+ boolean valueMatches = arrayContainsValue(sortedKeys);
+ if (valueMatches && !invertMatch || !valueMatches && invertMatch) {
+ try (final RowSet rowSet = rowSequence.asRowSet()) {
+ return rowSet.copy();
+ }
+ }
+
+ return RowSetFactory.empty();
+ }
+
+ private boolean arrayContainsValue(final Object[] sortedKeys) {
+ if (value == QueryConstants.NULL_DOUBLE && sortedKeys.length > 0
+ && (sortedKeys[0] == null || sortedKeys[0] == QueryConstants.NULL_DOUBLE_BOXED)) {
+ return true;
+ }
+ return Arrays.binarySearch(sortedKeys, value) >= 0;
+ }
}
final class StaticPageStore
extends RegionedPageStore.Static>
implements ColumnRegionDouble {
- public StaticPageStore(@NotNull final Parameters parameters, @NotNull final ColumnRegionDouble[] regions) {
- super(parameters, regions);
+ public StaticPageStore(
+ @NotNull final Parameters parameters,
+ @NotNull final ColumnRegionDouble[] regions,
+ @NotNull final ColumnLocation location) {
+ super(parameters, regions, location);
}
@Override
@@ -113,5 +175,15 @@ public double getDouble(final long elementIndex) {
public double getDouble(@NotNull final FillContext context, final long elementIndex) {
return lookupRegion(elementIndex).getDouble(context, elementIndex);
}
+
+ @Override
+ public WritableRowSet match(
+ final boolean invertMatch,
+ final boolean usePrev, boolean caseInsensitive,
+ @NotNull final RowSequence rowSequence,
+ final Object... sortedKeys) {
+ // where is the data stored here?
+ throw new UnsupportedOperationException("TODO NATE NOCOMMIT");
+ }
}
}
diff --git a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/ColumnRegionFloat.java b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/ColumnRegionFloat.java
index b8cc6489044..e0eb02b4a9a 100644
--- a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/ColumnRegionFloat.java
+++ b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/ColumnRegionFloat.java
@@ -8,13 +8,20 @@
*/
package io.deephaven.engine.table.impl.sources.regioned;
-import io.deephaven.chunk.attributes.Any;
import io.deephaven.chunk.ChunkType;
import io.deephaven.chunk.WritableChunk;
+import io.deephaven.chunk.attributes.Any;
+import io.deephaven.engine.rowset.RowSequence;
+import io.deephaven.engine.rowset.RowSet;
+import io.deephaven.engine.rowset.RowSetFactory;
+import io.deephaven.engine.rowset.WritableRowSet;
+import io.deephaven.engine.table.impl.locations.ColumnLocation;
import io.deephaven.util.QueryConstants;
import io.deephaven.util.annotations.FinalDefault;
import org.jetbrains.annotations.NotNull;
+import java.util.Arrays;
+
/**
* Column region interface for regions that support fetching primitive floats.
*/
@@ -63,6 +70,23 @@ private Null(final long pageMask) {
public float getFloat(final long elementIndex) {
return QueryConstants.NULL_FLOAT;
}
+
+ @Override
+ public WritableRowSet match(
+ final boolean invertMatch,
+ final boolean usePrev,
+ final boolean caseInsensitive,
+ @NotNull final RowSequence rowSequence,
+ final Object... sortedKeys) {
+ final boolean nullMatched = sortedKeys.length > 0
+ && (sortedKeys[0] == null || sortedKeys[0].equals(QueryConstants.NULL_FLOAT_BOXED));
+ if (nullMatched && !invertMatch || !nullMatched && invertMatch) {
+ try (final RowSet rowSet = rowSequence.asRowSet()) {
+ return rowSet.copy();
+ }
+ }
+ return RowSetFactory.empty();
+ }
}
final class Constant
@@ -87,14 +111,52 @@ public void fillChunkAppend(@NotNull final FillContext context, @NotNull final W
destination.asWritableFloatChunk().fillWithValue(offset, length, value);
destination.setSize(offset + length);
}
+
+ @Override
+ public ColumnLocation getLocation() {
+ return null;
+ }
+
+ @Override
+ public boolean supportsMatching() {
+ return true;
+ }
+
+ @Override
+ public WritableRowSet match(
+ final boolean invertMatch,
+ final boolean usePrev,
+ final boolean caseInsensitive,
+ @NotNull final RowSequence rowSequence,
+ final Object... sortedKeys) {
+ boolean valueMatches = arrayContainsValue(sortedKeys);
+ if (valueMatches && !invertMatch || !valueMatches && invertMatch) {
+ try (final RowSet rowSet = rowSequence.asRowSet()) {
+ return rowSet.copy();
+ }
+ }
+
+ return RowSetFactory.empty();
+ }
+
+ private boolean arrayContainsValue(final Object[] sortedKeys) {
+ if (value == QueryConstants.NULL_FLOAT && sortedKeys.length > 0
+ && (sortedKeys[0] == null || sortedKeys[0] == QueryConstants.NULL_FLOAT_BOXED)) {
+ return true;
+ }
+ return Arrays.binarySearch(sortedKeys, value) >= 0;
+ }
}
final class StaticPageStore
extends RegionedPageStore.Static>
implements ColumnRegionFloat {
- public StaticPageStore(@NotNull final Parameters parameters, @NotNull final ColumnRegionFloat[] regions) {
- super(parameters, regions);
+ public StaticPageStore(
+ @NotNull final Parameters parameters,
+ @NotNull final ColumnRegionFloat[] regions,
+ @NotNull final ColumnLocation location) {
+ super(parameters, regions, location);
}
@Override
@@ -113,5 +175,15 @@ public float getFloat(final long elementIndex) {
public float getFloat(@NotNull final FillContext context, final long elementIndex) {
return lookupRegion(elementIndex).getFloat(context, elementIndex);
}
+
+ @Override
+ public WritableRowSet match(
+ final boolean invertMatch,
+ final boolean usePrev, boolean caseInsensitive,
+ @NotNull final RowSequence rowSequence,
+ final Object... sortedKeys) {
+ // where is the data stored here?
+ throw new UnsupportedOperationException("TODO NATE NOCOMMIT");
+ }
}
}
diff --git a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/ColumnRegionInt.java b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/ColumnRegionInt.java
index 44e7863f418..b0d440a3bbb 100644
--- a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/ColumnRegionInt.java
+++ b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/ColumnRegionInt.java
@@ -8,13 +8,20 @@
*/
package io.deephaven.engine.table.impl.sources.regioned;
-import io.deephaven.chunk.attributes.Any;
import io.deephaven.chunk.ChunkType;
import io.deephaven.chunk.WritableChunk;
+import io.deephaven.chunk.attributes.Any;
+import io.deephaven.engine.rowset.RowSequence;
+import io.deephaven.engine.rowset.RowSet;
+import io.deephaven.engine.rowset.RowSetFactory;
+import io.deephaven.engine.rowset.WritableRowSet;
+import io.deephaven.engine.table.impl.locations.ColumnLocation;
import io.deephaven.util.QueryConstants;
import io.deephaven.util.annotations.FinalDefault;
import org.jetbrains.annotations.NotNull;
+import java.util.Arrays;
+
/**
* Column region interface for regions that support fetching primitive ints.
*/
@@ -63,6 +70,23 @@ private Null(final long pageMask) {
public int getInt(final long elementIndex) {
return QueryConstants.NULL_INT;
}
+
+ @Override
+ public WritableRowSet match(
+ final boolean invertMatch,
+ final boolean usePrev,
+ final boolean caseInsensitive,
+ @NotNull final RowSequence rowSequence,
+ final Object... sortedKeys) {
+ final boolean nullMatched = sortedKeys.length > 0
+ && (sortedKeys[0] == null || sortedKeys[0].equals(QueryConstants.NULL_INT_BOXED));
+ if (nullMatched && !invertMatch || !nullMatched && invertMatch) {
+ try (final RowSet rowSet = rowSequence.asRowSet()) {
+ return rowSet.copy();
+ }
+ }
+ return RowSetFactory.empty();
+ }
}
final class Constant
@@ -87,14 +111,52 @@ public void fillChunkAppend(@NotNull final FillContext context, @NotNull final W
destination.asWritableIntChunk().fillWithValue(offset, length, value);
destination.setSize(offset + length);
}
+
+ @Override
+ public ColumnLocation getLocation() {
+ return null;
+ }
+
+ @Override
+ public boolean supportsMatching() {
+ return true;
+ }
+
+ @Override
+ public WritableRowSet match(
+ final boolean invertMatch,
+ final boolean usePrev,
+ final boolean caseInsensitive,
+ @NotNull final RowSequence rowSequence,
+ final Object... sortedKeys) {
+ boolean valueMatches = arrayContainsValue(sortedKeys);
+ if (valueMatches && !invertMatch || !valueMatches && invertMatch) {
+ try (final RowSet rowSet = rowSequence.asRowSet()) {
+ return rowSet.copy();
+ }
+ }
+
+ return RowSetFactory.empty();
+ }
+
+ private boolean arrayContainsValue(final Object[] sortedKeys) {
+ if (value == QueryConstants.NULL_INT && sortedKeys.length > 0
+ && (sortedKeys[0] == null || sortedKeys[0] == QueryConstants.NULL_INT_BOXED)) {
+ return true;
+ }
+ return Arrays.binarySearch(sortedKeys, value) >= 0;
+ }
}
final class StaticPageStore
extends RegionedPageStore.Static>
implements ColumnRegionInt {
- public StaticPageStore(@NotNull final Parameters parameters, @NotNull final ColumnRegionInt[] regions) {
- super(parameters, regions);
+ public StaticPageStore(
+ @NotNull final Parameters parameters,
+ @NotNull final ColumnRegionInt[] regions,
+ @NotNull final ColumnLocation location) {
+ super(parameters, regions, location);
}
@Override
@@ -113,5 +175,15 @@ public int getInt(final long elementIndex) {
public int getInt(@NotNull final FillContext context, final long elementIndex) {
return lookupRegion(elementIndex).getInt(context, elementIndex);
}
+
+ @Override
+ public WritableRowSet match(
+ final boolean invertMatch,
+ final boolean usePrev, boolean caseInsensitive,
+ @NotNull final RowSequence rowSequence,
+ final Object... sortedKeys) {
+ // where is the data stored here?
+ throw new UnsupportedOperationException("TODO NATE NOCOMMIT");
+ }
}
}
diff --git a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/ColumnRegionLong.java b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/ColumnRegionLong.java
index a1cf5ee8856..3ca560ade8a 100644
--- a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/ColumnRegionLong.java
+++ b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/ColumnRegionLong.java
@@ -8,13 +8,20 @@
*/
package io.deephaven.engine.table.impl.sources.regioned;
-import io.deephaven.chunk.attributes.Any;
import io.deephaven.chunk.ChunkType;
import io.deephaven.chunk.WritableChunk;
+import io.deephaven.chunk.attributes.Any;
+import io.deephaven.engine.rowset.RowSequence;
+import io.deephaven.engine.rowset.RowSet;
+import io.deephaven.engine.rowset.RowSetFactory;
+import io.deephaven.engine.rowset.WritableRowSet;
+import io.deephaven.engine.table.impl.locations.ColumnLocation;
import io.deephaven.util.QueryConstants;
import io.deephaven.util.annotations.FinalDefault;
import org.jetbrains.annotations.NotNull;
+import java.util.Arrays;
+
/**
* Column region interface for regions that support fetching primitive longs.
*/
@@ -63,6 +70,23 @@ private Null(final long pageMask) {
public long getLong(final long elementIndex) {
return QueryConstants.NULL_LONG;
}
+
+ @Override
+ public WritableRowSet match(
+ final boolean invertMatch,
+ final boolean usePrev,
+ final boolean caseInsensitive,
+ @NotNull final RowSequence rowSequence,
+ final Object... sortedKeys) {
+ final boolean nullMatched = sortedKeys.length > 0
+ && (sortedKeys[0] == null || sortedKeys[0].equals(QueryConstants.NULL_LONG_BOXED));
+ if (nullMatched && !invertMatch || !nullMatched && invertMatch) {
+ try (final RowSet rowSet = rowSequence.asRowSet()) {
+ return rowSet.copy();
+ }
+ }
+ return RowSetFactory.empty();
+ }
}
final class Constant
@@ -87,14 +111,52 @@ public void fillChunkAppend(@NotNull final FillContext context, @NotNull final W
destination.asWritableLongChunk().fillWithValue(offset, length, value);
destination.setSize(offset + length);
}
+
+ @Override
+ public ColumnLocation getLocation() {
+ return null;
+ }
+
+ @Override
+ public boolean supportsMatching() {
+ return true;
+ }
+
+ @Override
+ public WritableRowSet match(
+ final boolean invertMatch,
+ final boolean usePrev,
+ final boolean caseInsensitive,
+ @NotNull final RowSequence rowSequence,
+ final Object... sortedKeys) {
+ boolean valueMatches = arrayContainsValue(sortedKeys);
+ if (valueMatches && !invertMatch || !valueMatches && invertMatch) {
+ try (final RowSet rowSet = rowSequence.asRowSet()) {
+ return rowSet.copy();
+ }
+ }
+
+ return RowSetFactory.empty();
+ }
+
+ private boolean arrayContainsValue(final Object[] sortedKeys) {
+ if (value == QueryConstants.NULL_LONG && sortedKeys.length > 0
+ && (sortedKeys[0] == null || sortedKeys[0] == QueryConstants.NULL_LONG_BOXED)) {
+ return true;
+ }
+ return Arrays.binarySearch(sortedKeys, value) >= 0;
+ }
}
final class StaticPageStore
extends RegionedPageStore.Static>
implements ColumnRegionLong {
- public StaticPageStore(@NotNull final Parameters parameters, @NotNull final ColumnRegionLong[] regions) {
- super(parameters, regions);
+ public StaticPageStore(
+ @NotNull final Parameters parameters,
+ @NotNull final ColumnRegionLong[] regions,
+ @NotNull final ColumnLocation location) {
+ super(parameters, regions, location);
}
@Override
@@ -113,5 +175,15 @@ public long getLong(final long elementIndex) {
public long getLong(@NotNull final FillContext context, final long elementIndex) {
return lookupRegion(elementIndex).getLong(context, elementIndex);
}
+
+ @Override
+ public WritableRowSet match(
+ final boolean invertMatch,
+ final boolean usePrev, boolean caseInsensitive,
+ @NotNull final RowSequence rowSequence,
+ final Object... sortedKeys) {
+ // where is the data stored here?
+ throw new UnsupportedOperationException("TODO NATE NOCOMMIT");
+ }
}
}
diff --git a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/ColumnRegionObject.java b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/ColumnRegionObject.java
index 606431ae788..083ed278b2c 100644
--- a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/ColumnRegionObject.java
+++ b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/ColumnRegionObject.java
@@ -4,17 +4,17 @@
package io.deephaven.engine.table.impl.sources.regioned;
import io.deephaven.chunk.attributes.Any;
+import io.deephaven.engine.rowset.*;
import io.deephaven.engine.table.impl.chunkattributes.DictionaryKeys;
import io.deephaven.chunk.ChunkType;
import io.deephaven.chunk.WritableChunk;
import io.deephaven.chunk.WritableLongChunk;
import io.deephaven.engine.page.Page;
-import io.deephaven.engine.rowset.RowSequence;
-import io.deephaven.engine.rowset.RowSet;
-import io.deephaven.engine.rowset.RowSetBuilderSequential;
+import io.deephaven.engine.table.impl.locations.ColumnLocation;
import io.deephaven.util.annotations.FinalDefault;
import org.jetbrains.annotations.NotNull;
+import java.util.Arrays;
import java.util.stream.IntStream;
import static io.deephaven.util.QueryConstants.NULL_LONG;
@@ -161,6 +161,22 @@ public boolean gatherDictionaryValuesRowSet(@NotNull final RowSet.SearchIterator
public ColumnRegionLong getDictionaryKeysRegion() {
return dictionaryKeysRegion == null ? dictionaryKeysRegion = ColumnRegionLong.createNull(mask()) : dictionaryKeysRegion;
}
+
+ @Override
+ public WritableRowSet match(
+ final boolean invertMatch,
+ final boolean usePrev,
+ final boolean caseInsensitive,
+ @NotNull final RowSequence rowSequence,
+ final Object... sortedKeys) {
+ final boolean nullMatched = sortedKeys.length > 0 && sortedKeys[0] == null;
+ if (nullMatched && !invertMatch || !nullMatched && invertMatch) {
+ try (final RowSet rowSet = rowSequence.asRowSet()) {
+ return rowSet.copy();
+ }
+ }
+ return RowSetFactory.empty();
+ }
}
static ColumnRegionLong createConstantDictionaryKeysRegion(final long pageMask) {
@@ -212,6 +228,40 @@ public boolean gatherDictionaryValuesRowSet(@NotNull final RowSet.SearchIterator
public ColumnRegionLong getDictionaryKeysRegion() {
return dictionaryKeysRegion == null ? dictionaryKeysRegion = createConstantDictionaryKeysRegion(mask()) : dictionaryKeysRegion;
}
+
+ @Override
+ public ColumnLocation getLocation() {
+ return null;
+ }
+
+ @Override
+ public boolean supportsMatching() {
+ return true;
+ }
+
+ @Override
+ public WritableRowSet match(
+ final boolean invertMatch,
+ final boolean usePrev,
+ final boolean caseInsensitive,
+ @NotNull final RowSequence rowSequence,
+ final Object... sortedKeys) {
+ final boolean valueMatches = arrayContainsValue(sortedKeys);
+ if (valueMatches && !invertMatch || !valueMatches && invertMatch) {
+ try (final RowSet rowSet = rowSequence.asRowSet()) {
+ return rowSet.copy();
+ }
+ }
+
+ return RowSetFactory.empty();
+ }
+
+ private boolean arrayContainsValue(final Object[] sortedKeys) {
+ if (value == null && sortedKeys.length > 0 && sortedKeys[0] == null) {
+ return true;
+ }
+ return Arrays.asList(sortedKeys).contains(value);
+ }
}
final class StaticPageStore
@@ -221,8 +271,11 @@ final class StaticPageStore
private ColumnRegionLong dictionaryKeysRegion;
private ColumnRegionObject dictionaryValuesRegion;
- public StaticPageStore(@NotNull final Parameters parameters, @NotNull final ColumnRegionObject[] regions) {
- super(parameters, regions);
+ public StaticPageStore(
+ @NotNull final Parameters parameters,
+ @NotNull final ColumnRegionObject[] regions,
+ @NotNull final ColumnLocation location) {
+ super(parameters, regions, location);
}
@Override
@@ -267,14 +320,16 @@ public boolean gatherDictionaryValuesRowSet(@NotNull final RowSet.SearchIterator
@Override
public ColumnRegionLong getDictionaryKeysRegion() {
return dictionaryKeysRegion == null
- ? dictionaryKeysRegion = new ColumnRegionLong.StaticPageStore<>(parameters(), mapRegionsToDictionaryKeys())
+ ? dictionaryKeysRegion = new ColumnRegionLong.StaticPageStore<>(
+ parameters(), mapRegionsToDictionaryKeys(), getLocation())
: dictionaryKeysRegion;
}
@Override
public ColumnRegionObject getDictionaryValuesRegion() {
return dictionaryValuesRegion == null
- ? dictionaryValuesRegion = new ColumnRegionObject.StaticPageStore<>(parameters(), mapRegionsToDictionaryValues())
+ ? dictionaryValuesRegion = new ColumnRegionObject.StaticPageStore<>(
+ parameters(), mapRegionsToDictionaryValues(), getLocation())
: dictionaryValuesRegion;
}
@@ -291,6 +346,16 @@ private ColumnRegionObject[] mapRegionsToDictionaryValues() {
.mapToObj(ri -> getRegion(ri).getDictionaryValuesRegion())
.toArray(ColumnRegionObject[]::new);
}
+
+ @Override
+ public WritableRowSet match(
+ final boolean invertMatch,
+ final boolean usePrev, boolean caseInsensitive,
+ @NotNull final RowSequence rowSequence,
+ final Object... sortedKeys) {
+ // where is the data stored here?
+ throw new UnsupportedOperationException("TODO NATE NOCOMMIT");
+ }
}
final class DictionaryKeysWrapper implements ColumnRegionLong, Page.WithDefaults {
@@ -350,5 +415,25 @@ public void fillChunkAppend(@NotNull final FillContext context, @NotNull final W
}
}
}
+
+ @Override
+ public boolean supportsMatching() {
+ return wrapped.supportsMatching();
+ }
+
+ @Override
+ public WritableRowSet match(
+ final boolean invertMatch,
+ final boolean usePrev,
+ final boolean caseInsensitive,
+ @NotNull final RowSequence rowSequence,
+ final Object... sortedKeys) {
+ return wrapped.match(invertMatch, usePrev, caseInsensitive, rowSequence, sortedKeys);
+ }
+
+ @Override
+ public ColumnLocation getLocation() {
+ return wrapped.getLocation();
+ }
}
}
diff --git a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/ColumnRegionReferencing.java b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/ColumnRegionReferencing.java
index 688b52ca829..9b2063f6eca 100644
--- a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/ColumnRegionReferencing.java
+++ b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/ColumnRegionReferencing.java
@@ -8,6 +8,7 @@
import io.deephaven.chunk.ChunkType;
import io.deephaven.chunk.WritableChunk;
import io.deephaven.engine.rowset.RowSequence;
+import io.deephaven.engine.rowset.WritableRowSet;
import io.deephaven.util.annotations.FinalDefault;
import org.jetbrains.annotations.NotNull;
@@ -33,6 +34,15 @@ void convertRegion(
@NotNull WritableChunk super ATTR> destination,
@NotNull Chunk extends ATTR> source,
@NotNull RowSequence rowSequence);
+
+ /**
+ * Convert an array of values in the region type to an array of values in boxed native format.
+ *
+ * @param values the values to convert
+ * @return a new array of converted values
+ */
+ @NotNull
+ Object[] convertArray(@NotNull Object[] values);
}
class Null>
@@ -51,5 +61,15 @@ public Null(REFERENCED_COLUMN_REGION nullReferencedColumnRegion) {
public REFERENCED_COLUMN_REGION getReferencedRegion() {
return nullReferencedColumnRegion;
}
+
+ @Override
+ public WritableRowSet match(
+ final boolean invertMatch,
+ final boolean usePrev,
+ final boolean caseInsensitive,
+ @NotNull final RowSequence rowSequence,
+ final Object... sortedKeys) {
+ return nullReferencedColumnRegion.match(invertMatch, usePrev, caseInsensitive, rowSequence, sortedKeys);
+ }
}
}
diff --git a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/ColumnRegionReferencingImpl.java b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/ColumnRegionReferencingImpl.java
index 4b8b10bbbcd..ba927e31ee7 100644
--- a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/ColumnRegionReferencingImpl.java
+++ b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/ColumnRegionReferencingImpl.java
@@ -4,13 +4,20 @@
package io.deephaven.engine.table.impl.sources.regioned;
import io.deephaven.chunk.attributes.Any;
+import io.deephaven.chunk.attributes.Values;
+import io.deephaven.engine.rowset.RowSet;
+import io.deephaven.engine.rowset.WritableRowSet;
import io.deephaven.engine.table.ChunkSource;
import io.deephaven.engine.table.GetContextMaker;
import io.deephaven.engine.table.SharedContext;
import io.deephaven.chunk.WritableChunk;
import io.deephaven.engine.page.Page;
import io.deephaven.engine.rowset.RowSequence;
+import io.deephaven.engine.table.impl.chunkfilter.ChunkFilter;
+import io.deephaven.engine.table.impl.chunkfilter.ChunkMatchFilterFactory;
+import io.deephaven.engine.table.impl.locations.ColumnLocation;
import org.jetbrains.annotations.NotNull;
+import org.jetbrains.annotations.Nullable;
import javax.annotation.OverridingMethodsMustInvokeSuper;
@@ -18,9 +25,14 @@ public class ColumnRegionReferencingImpl, Page.WithDefaults {
private final REFERENCED_COLUMN_REGION referencedColumnRegion;
+ @NotNull
+ private final Converter converter;
- public ColumnRegionReferencingImpl(@NotNull final REFERENCED_COLUMN_REGION referencedColumnRegion) {
+ public ColumnRegionReferencingImpl(
+ @NotNull final REFERENCED_COLUMN_REGION referencedColumnRegion,
+ @NotNull final Converter converter) {
this.referencedColumnRegion = referencedColumnRegion;
+ this.converter = converter;
}
@Override
@@ -42,10 +54,16 @@ public long mask() {
@Override
public void fillChunkAppend(@NotNull ChunkSource.FillContext context,
@NotNull WritableChunk super ATTR> destination, @NotNull RowSequence rowSequence) {
- FillContext.converter(context).convertRegion(destination,
+ converter.convertRegion(destination,
referencedColumnRegion.getChunk(FillContext.nativeGetContext(context), rowSequence), rowSequence);
}
+ @Override
+ public ChunkSource.FillContext makeFillContext(
+ final int chunkCapacity, @Nullable final SharedContext sharedContext) {
+ return new FillContext(referencedColumnRegion, chunkCapacity, sharedContext);
+ }
+
@Override
@OverridingMethodsMustInvokeSuper
public void releaseCachedResources() {
@@ -53,25 +71,48 @@ public void releaseCachedResources() {
referencedColumnRegion.releaseCachedResources();
}
+ @Override
+ public ColumnLocation getLocation() {
+ return referencedColumnRegion.getLocation();
+ }
+
+ @Override
+ public boolean supportsMatching() {
+ return true;
+ }
+
+ @Override
+ public WritableRowSet match(
+ final boolean invertMatch,
+ final boolean usePrev,
+ final boolean caseInsensitive,
+ @NotNull final RowSequence rowSequence,
+ final Object... sortedKeys) {
+ final Object[] nativeKeys = converter.convertArray(sortedKeys);
+ try (final RowSet rows = rowSequence.asRowSet()) {
+ // TODO NATE NOCOMMIT: getNativeType? or pushdown?
+ // noinspection unchecked
+ return ChunkFilter.applyChunkFilter(rows, (ChunkSource extends Values>) referencedColumnRegion, false,
+ ChunkMatchFilterFactory.getChunkFilter(
+ null, caseInsensitive, invertMatch, nativeKeys));
+ // referencedColumnRegion.getNativeType(), caseInsensitive, invertMatch, nativeKeys));
+ }
+ }
+
static class FillContext implements ChunkSource.FillContext {
private final ChunkSource.GetContext nativeGetContext;
- private final Converter converter;
- FillContext(GetContextMaker getContextMaker, Converter converter, int chunkCapacity,
- SharedContext sharedContext) {
- this.converter = converter;
+ FillContext(
+ @NotNull final GetContextMaker getContextMaker,
+ int chunkCapacity,
+ @Nullable final SharedContext sharedContext) {
this.nativeGetContext = getContextMaker.makeGetContext(chunkCapacity, sharedContext);
}
- static ChunkSource.GetContext nativeGetContext(ChunkSource.FillContext context) {
+ static ChunkSource.GetContext nativeGetContext(@NotNull final ChunkSource.FillContext context) {
return ((FillContext>) context).nativeGetContext;
}
- static Converter converter(ChunkSource.FillContext context) {
- // noinspection unchecked
- return ((FillContext) context).converter;
- }
-
@Override
public void close() {
nativeGetContext.close();
diff --git a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/ColumnRegionShort.java b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/ColumnRegionShort.java
index 588f34eeb7c..fa634b528df 100644
--- a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/ColumnRegionShort.java
+++ b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/ColumnRegionShort.java
@@ -8,13 +8,20 @@
*/
package io.deephaven.engine.table.impl.sources.regioned;
-import io.deephaven.chunk.attributes.Any;
import io.deephaven.chunk.ChunkType;
import io.deephaven.chunk.WritableChunk;
+import io.deephaven.chunk.attributes.Any;
+import io.deephaven.engine.rowset.RowSequence;
+import io.deephaven.engine.rowset.RowSet;
+import io.deephaven.engine.rowset.RowSetFactory;
+import io.deephaven.engine.rowset.WritableRowSet;
+import io.deephaven.engine.table.impl.locations.ColumnLocation;
import io.deephaven.util.QueryConstants;
import io.deephaven.util.annotations.FinalDefault;
import org.jetbrains.annotations.NotNull;
+import java.util.Arrays;
+
/**
* Column region interface for regions that support fetching primitive shorts.
*/
@@ -63,6 +70,23 @@ private Null(final long pageMask) {
public short getShort(final long elementIndex) {
return QueryConstants.NULL_SHORT;
}
+
+ @Override
+ public WritableRowSet match(
+ final boolean invertMatch,
+ final boolean usePrev,
+ final boolean caseInsensitive,
+ @NotNull final RowSequence rowSequence,
+ final Object... sortedKeys) {
+ final boolean nullMatched = sortedKeys.length > 0
+ && (sortedKeys[0] == null || sortedKeys[0].equals(QueryConstants.NULL_SHORT_BOXED));
+ if (nullMatched && !invertMatch || !nullMatched && invertMatch) {
+ try (final RowSet rowSet = rowSequence.asRowSet()) {
+ return rowSet.copy();
+ }
+ }
+ return RowSetFactory.empty();
+ }
}
final class Constant
@@ -87,14 +111,52 @@ public void fillChunkAppend(@NotNull final FillContext context, @NotNull final W
destination.asWritableShortChunk().fillWithValue(offset, length, value);
destination.setSize(offset + length);
}
+
+ @Override
+ public ColumnLocation getLocation() {
+ return null;
+ }
+
+ @Override
+ public boolean supportsMatching() {
+ return true;
+ }
+
+ @Override
+ public WritableRowSet match(
+ final boolean invertMatch,
+ final boolean usePrev,
+ final boolean caseInsensitive,
+ @NotNull final RowSequence rowSequence,
+ final Object... sortedKeys) {
+ boolean valueMatches = arrayContainsValue(sortedKeys);
+ if (valueMatches && !invertMatch || !valueMatches && invertMatch) {
+ try (final RowSet rowSet = rowSequence.asRowSet()) {
+ return rowSet.copy();
+ }
+ }
+
+ return RowSetFactory.empty();
+ }
+
+ private boolean arrayContainsValue(final Object[] sortedKeys) {
+ if (value == QueryConstants.NULL_SHORT && sortedKeys.length > 0
+ && (sortedKeys[0] == null || sortedKeys[0] == QueryConstants.NULL_SHORT_BOXED)) {
+ return true;
+ }
+ return Arrays.binarySearch(sortedKeys, value) >= 0;
+ }
}
final class StaticPageStore
extends RegionedPageStore.Static>
implements ColumnRegionShort {
- public StaticPageStore(@NotNull final Parameters parameters, @NotNull final ColumnRegionShort[] regions) {
- super(parameters, regions);
+ public StaticPageStore(
+ @NotNull final Parameters parameters,
+ @NotNull final ColumnRegionShort[] regions,
+ @NotNull final ColumnLocation location) {
+ super(parameters, regions, location);
}
@Override
@@ -113,5 +175,15 @@ public short getShort(final long elementIndex) {
public short getShort(@NotNull final FillContext context, final long elementIndex) {
return lookupRegion(elementIndex).getShort(context, elementIndex);
}
+
+ @Override
+ public WritableRowSet match(
+ final boolean invertMatch,
+ final boolean usePrev, boolean caseInsensitive,
+ @NotNull final RowSequence rowSequence,
+ final Object... sortedKeys) {
+ // where is the data stored here?
+ throw new UnsupportedOperationException("TODO NATE NOCOMMIT");
+ }
}
}
diff --git a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/DeferredColumnRegionBase.java b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/DeferredColumnRegionBase.java
index 37258661c0e..cd35f0900df 100644
--- a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/DeferredColumnRegionBase.java
+++ b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/DeferredColumnRegionBase.java
@@ -5,9 +5,11 @@
import io.deephaven.base.verify.Require;
import io.deephaven.chunk.attributes.Any;
+import io.deephaven.engine.rowset.WritableRowSet;
import io.deephaven.engine.table.SharedContext;
import io.deephaven.chunk.*;
import io.deephaven.engine.rowset.RowSequence;
+import io.deephaven.engine.table.impl.locations.ColumnLocation;
import org.jetbrains.annotations.NotNull;
import javax.annotation.OverridingMethodsMustInvokeSuper;
@@ -77,8 +79,10 @@ public ChunkType getChunkType() {
}
@Override
- public void fillChunk(@NotNull FillContext context, @NotNull WritableChunk super ATTR> destination,
- @NotNull RowSequence rowSequence) {
+ public void fillChunk(
+ @NotNull final FillContext context,
+ @NotNull final WritableChunk super ATTR> destination,
+ @NotNull final RowSequence rowSequence) {
getResultRegion().fillChunk(context, destination, rowSequence);
}
@@ -91,22 +95,42 @@ public void fillChunkAppend(
}
@Override
- public Chunk extends ATTR> getChunk(@NotNull GetContext context, @NotNull RowSequence rowSequence) {
+ public Chunk extends ATTR> getChunk(@NotNull final GetContext context, @NotNull final RowSequence rowSequence) {
return getResultRegion().getChunk(context, rowSequence);
}
@Override
- public Chunk extends ATTR> getChunk(@NotNull GetContext context, long firstKey, long lastKey) {
+ public Chunk extends ATTR> getChunk(@NotNull final GetContext context, final long firstKey, final long lastKey) {
return getResultRegion().getChunk(context, firstKey, lastKey);
}
@Override
- public FillContext makeFillContext(int chunkCapacity, SharedContext sharedContext) {
+ public FillContext makeFillContext(final int chunkCapacity, final SharedContext sharedContext) {
return getResultRegion().makeFillContext(chunkCapacity, sharedContext);
}
@Override
- public GetContext makeGetContext(int chunkCapacity, SharedContext sharedContext) {
+ public GetContext makeGetContext(final int chunkCapacity, final SharedContext sharedContext) {
return getResultRegion().makeGetContext(chunkCapacity, sharedContext);
}
+
+ @Override
+ public ColumnLocation getLocation() {
+ return getResultRegion().getLocation();
+ }
+
+ @Override
+ public boolean supportsMatching() {
+ return getResultRegion().supportsMatching();
+ }
+
+ @Override
+ public WritableRowSet match(
+ final boolean invertMatch,
+ final boolean usePrev,
+ final boolean caseInsensitive,
+ @NotNull final RowSequence rowSequence,
+ final Object... sortedKeys) {
+ return getResultRegion().match(invertMatch, usePrev, caseInsensitive, rowSequence, sortedKeys);
+ }
}
diff --git a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/DeferredColumnRegionReferencing.java b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/DeferredColumnRegionReferencing.java
deleted file mode 100644
index 0ea34389d48..00000000000
--- a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/DeferredColumnRegionReferencing.java
+++ /dev/null
@@ -1,30 +0,0 @@
-/**
- * Copyright (c) 2016-2022 Deephaven Data Labs and Patent Pending
- */
-package io.deephaven.engine.table.impl.sources.regioned;
-
-import io.deephaven.chunk.attributes.Any;
-import org.jetbrains.annotations.NotNull;
-
-import java.util.function.Supplier;
-
-/**
- * {@link ColumnRegionReferencing} implementation for deferred regions, i.e. regions that will be properly constructed
- * on first access.
- */
-public class DeferredColumnRegionReferencing>
- extends DeferredColumnRegionBase>
- implements ColumnRegionReferencing {
-
- DeferredColumnRegionReferencing(final long pageMask,
- @NotNull Supplier> resultRegionFactory) {
- super(pageMask, resultRegionFactory);
- }
-
- @NotNull
- @Override
- public REFERENCED_COLUMN_REGION getReferencedRegion() {
- return getResultRegion().getReferencedRegion();
- }
-
-}
diff --git a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/MakeRegion.java b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/MakeRegion.java
index bf7683325f4..45a64edb5f3 100644
--- a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/MakeRegion.java
+++ b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/MakeRegion.java
@@ -6,6 +6,7 @@
import io.deephaven.chunk.attributes.Values;
import io.deephaven.engine.table.ColumnDefinition;
import io.deephaven.engine.table.impl.locations.ColumnLocation;
+import io.deephaven.engine.table.impl.sources.regioned.instructions.SourceTableColumnInstructions;
import org.jetbrains.annotations.NotNull;
import org.jetbrains.annotations.Nullable;
@@ -17,6 +18,7 @@ public interface MakeRegion columnDefinition,
+ REGION_TYPE makeRegion(
+ @NotNull ColumnDefinition> columnDefinition,
@NotNull ColumnLocation columnLocation,
+ @NotNull SourceTableColumnInstructions instructions,
int regionIndex);
}
diff --git a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSource.java b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSource.java
index ce0afa78d5f..f76b73715a7 100644
--- a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSource.java
+++ b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSource.java
@@ -9,6 +9,8 @@
import io.deephaven.engine.table.impl.sources.DeferredGroupingColumnSource;
import io.deephaven.engine.table.impl.ImmutableColumnSource;
import io.deephaven.engine.rowset.RowSet;
+import io.deephaven.engine.rowset.TrackingWritableRowSet;
+import io.deephaven.engine.table.impl.sources.regioned.instructions.SourceTableColumnInstructions;
import io.deephaven.util.annotations.VisibleForTesting;
import org.jetbrains.annotations.NotNull;
@@ -106,10 +108,13 @@ static long getRowKey(final int regionIndex, final long regionOffset) {
*
* @param columnDefinition The column definition for this column source (potentially varies by region)
* @param columnLocation The column location for the region being added
+ * @param instructions The instructions for the region being added
* @return The index assigned to the added region
*/
- int addRegion(@NotNull final ColumnDefinition> columnDefinition,
- @NotNull final ColumnLocation columnLocation);
+ int addRegion(
+ @NotNull final ColumnDefinition> columnDefinition,
+ @NotNull final ColumnLocation columnLocation,
+ @NotNull final SourceTableColumnInstructions instructions);
/**
* Invalidate the specified region. An invalidated region will throw an exception on any read attempt if it cannot
@@ -118,4 +123,9 @@ int addRegion(@NotNull final ColumnDefinition> columnDefinition,
* @param regionIndex the region to invalidate
*/
void invalidateRegion(int regionIndex);
+
+ /**
+ * Disable the use of grouping for any operation of this source.
+ */
+ void disableGrouping();
}
diff --git a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceArray.java b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceArray.java
index b43d4fdbfb6..cbbaf71112c 100644
--- a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceArray.java
+++ b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceArray.java
@@ -10,6 +10,7 @@
import io.deephaven.chunk.attributes.Values;
import io.deephaven.engine.table.ColumnDefinition;
import io.deephaven.engine.table.impl.locations.ColumnLocation;
+import io.deephaven.engine.table.impl.sources.regioned.instructions.SourceTableColumnInstructions;
import io.deephaven.util.annotations.TestUseOnly;
import org.jetbrains.annotations.NotNull;
import org.jetbrains.annotations.Nullable;
@@ -80,19 +81,21 @@ private static > REG
@Override
@OverridingMethodsMustInvokeSuper
- public synchronized int addRegion(@NotNull final ColumnDefinition> columnDefinition,
- @NotNull final ColumnLocation columnLocation) {
+ public synchronized int addRegion(
+ @NotNull final ColumnDefinition> columnDefinition,
+ @NotNull final ColumnLocation columnLocation,
+ @NotNull final SourceTableColumnInstructions instructions) {
maybeExtendRegions();
final int regionIndex = regionCount;
- regions[regionIndex] = makeDeferred.make(PARAMETERS.regionMask,
- () -> updateRegion(regionIndex, makeRegion(columnDefinition, columnLocation, regionIndex)));
+ regions[regionIndex] = makeDeferred.make(PARAMETERS.regionMask, () -> updateRegion(regionIndex,
+ makeRegion(columnDefinition, columnLocation, instructions, regionIndex)));
return regionCount++;
}
/**
*
* Add a pre-constructed region without going through the abstract factory method
- * {@link #makeRegion(ColumnDefinition, ColumnLocation, int)}.
+ * {@link #makeRegion(ColumnDefinition, ColumnLocation, SourceTableColumnInstructions, int)}.
*
* This method is for unit testing purposes only!
*
diff --git a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceBase.java b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceBase.java
index 7ac1e19e170..a9205fc9e7c 100644
--- a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceBase.java
+++ b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceBase.java
@@ -5,19 +5,43 @@
import io.deephaven.base.verify.Assert;
import io.deephaven.chunk.attributes.Values;
+import io.deephaven.configuration.Configuration;
+import io.deephaven.engine.rowset.*;
import io.deephaven.engine.table.impl.sources.AbstractDeferredGroupingColumnSource;
import io.deephaven.chunk.WritableChunk;
-import io.deephaven.engine.rowset.RowSequence;
import io.deephaven.util.annotations.TestUseOnly;
+import io.deephaven.util.compare.ObjectComparisons;
import org.jetbrains.annotations.NotNull;
import org.jetbrains.annotations.Nullable;
+import java.util.ArrayList;
+import java.util.Arrays;
+import java.util.List;
+
/**
* Partial implementation of {@link RegionedColumnSource} for array-backed and delegating implementations to extend.
*/
-abstract class RegionedColumnSourceBase>
+public abstract class RegionedColumnSourceBase>
extends AbstractDeferredGroupingColumnSource
implements RegionedPageStore, RegionedColumnSource {
+ public static boolean USE_PARALLEL = Configuration.getInstance().getBooleanForClassWithDefault(
+ RegionedColumnSourceBase.class, "useParallelMatchFilter", true);
+ public static boolean USE_PUSHDOWN = Configuration.getInstance().getBooleanForClassWithDefault(
+ RegionedColumnSourceBase.class, "pushDownMatchFilter", true);
+
+ private static class MatchParams {
+ final ColumnRegion extends Values> region;
+ final RowSequence relevantKeys;
+ final long regionFirstkey;
+
+ private MatchParams(ColumnRegion extends Values> region, RowSequence relevantKeys, long regionFirstKey) {
+ this.region = region;
+ this.relevantKeys = relevantKeys;
+ this.regionFirstkey = regionFirstKey;
+ }
+ }
+
+ private volatile boolean groupingEnabled = true;
static final Parameters PARAMETERS;
static {
@@ -42,6 +66,11 @@ public void invalidateRegion(final int regionIndex) {
getRegion(regionIndex).invalidate();
}
+ @Override
+ public void disableGrouping() {
+ groupingEnabled = false;
+ }
+
@Override
public final Parameters parameters() {
return PARAMETERS;
@@ -83,4 +112,108 @@ public void fillPrevChunk(@NotNull FillContext context,
*/
@NotNull
abstract REGION_TYPE getNullRegion();
+
+ @Override
+ public WritableRowSet match(
+ final boolean invertMatch,
+ final boolean usePrev,
+ final boolean caseInsensitive,
+ @NotNull final RowSet rowSet,
+ final Object... keys) {
+ if (!USE_PUSHDOWN) {
+ return super.match(invertMatch, usePrev, caseInsensitive, rowSet, keys);
+ }
+
+ Arrays.sort(keys, ObjectComparisons::compare);
+ WritableRowSet result;
+ if (!USE_PARALLEL) {
+ result = doSequentialMatch(invertMatch, caseInsensitive, rowSet, keys);
+ } else {
+ result = doParallelMatch(invertMatch, caseInsensitive, rowSet, keys);
+ }
+
+ result.retain(rowSet);
+ return result;
+ }
+
+ private WritableRowSet doParallelMatch(
+ final boolean invertMatch,
+ final boolean caseInsensitive,
+ @NotNull final RowSet rowSet,
+ final Object... keys) {
+ // First find all the relevant regions for this set
+ final List relevantRegions = new ArrayList<>();
+ try (final RowSequence.Iterator rowSequenceIterator = rowSet.getRowSequenceIterator()) {
+ for (int regionIdx = 0; regionIdx < getRegionCount(); regionIdx++) {
+ final ColumnRegion extends Values> region = getRegion(regionIdx);
+ final long firstRegionIndex = region.firstRow(regionIdx);
+ final long lastRegionIndex = region.maxRow(regionIdx);
+ if (rowSequenceIterator.advance(firstRegionIndex)
+ && rowSequenceIterator.peekNextKey() <= lastRegionIndex) {
+ final RowSequence nextOrderedKeysThrough =
+ rowSequenceIterator.getNextRowSequenceThrough(lastRegionIndex);
+ relevantRegions.add(new MatchParams(region, nextOrderedKeysThrough.asRowSet(), firstRegionIndex));
+ }
+ }
+
+ // Now filter them all in parallel
+ final WritableRowSet result;
+ if (relevantRegions.size() == 1) {
+ final MatchParams p = relevantRegions.get(0);
+ result = matchWithinRegion(p.region,
+ p.relevantKeys,
+ caseInsensitive,
+ invertMatch,
+ keys).copy();
+ } else {
+ // TODO (deephaven-core#3851): parallelize; use disk backed deferred grouping pool if ported
+ final RowSetBuilderRandom resultBuilder = RowSetFactory.builderRandom();
+ relevantRegions.stream()
+ .map(p -> matchWithinRegion(p.region,
+ p.relevantKeys,
+ caseInsensitive,
+ invertMatch,
+ keys))
+ .forEach(resultBuilder::addRowSet);
+ result = resultBuilder.build();
+ }
+ return result;
+ }
+ }
+
+ private WritableRowSet doSequentialMatch(
+ final boolean invertMatch,
+ final boolean caseInsensitive,
+ @NotNull final RowSet startingRowSet,
+ final Object... keys) {
+ RowSetBuilderSequential resultBuilder = RowSetFactory.builderSequential();
+ try (final RowSequence.Iterator okIt = startingRowSet.getRowSequenceIterator()) {
+ for (int regionIdx = 0; regionIdx < getRegionCount(); regionIdx++) {
+ final ColumnRegion extends Values> region = getRegion(regionIdx);
+ final long firstRegionIndex = region.firstRow(regionIdx);
+ final long lastRegionIndex = region.maxRow(regionIdx);
+ if (okIt.advance(firstRegionIndex) && okIt.peekNextKey() <= lastRegionIndex) {
+ final RowSequence relevantRegionRows = okIt.getNextRowSequenceThrough(lastRegionIndex);
+ RowSet regionMatch = matchWithinRegion(region,
+ relevantRegionRows,
+ caseInsensitive,
+ invertMatch,
+ keys);
+ resultBuilder.appendRowSequence(regionMatch);
+ }
+ }
+ }
+
+ return resultBuilder.build();
+ }
+
+ private RowSet matchWithinRegion(
+ @NotNull final ColumnRegion extends Values> region,
+ @NotNull final RowSequence rowsToFilter,
+ final boolean caseInsensitive,
+ final boolean invertMatch,
+ @NotNull final Object[] keys) {
+ // TODO: ensure keys are sorted before parllelizing
+ return region.match(invertMatch, false, caseInsensitive, rowsToFilter, keys);
+ }
}
diff --git a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceBoolean.java b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceBoolean.java
index 7aea30c3742..3ad345523b5 100644
--- a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceBoolean.java
+++ b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceBoolean.java
@@ -47,4 +47,16 @@ public Boolean get(long rowKey) {
return rowKey == RowSequence.NULL_ROW_KEY ? null :
BooleanUtils.byteAsBoolean(getNativeSource().lookupRegion(rowKey).getByte(rowKey));
}
+
+ @Override
+ @NotNull
+ public Object[] convertArray(@NotNull final Object[] keys) {
+ final Object[] convertedValues = new Object[keys.length];
+ for(int ii = 0; ii < keys.length; ii++) {
+ if(keys[ii] == null || keys[ii] instanceof Boolean) {
+ convertedValues[ii] = BooleanUtils.booleanAsByte((Boolean)keys[ii]);
+ }
+ }
+ return convertedValues;
+ }
}
diff --git a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceByte.java b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceByte.java
index d617073347a..e60c594afa0 100644
--- a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceByte.java
+++ b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceByte.java
@@ -17,6 +17,7 @@
import io.deephaven.engine.table.impl.locations.TableLocationKey;
import io.deephaven.engine.table.impl.ColumnSourceGetDefaults;
import io.deephaven.chunk.attributes.Values;
+import io.deephaven.engine.table.impl.sources.regioned.instructions.SourceTableColumnInstructions;
import org.jetbrains.annotations.NotNull;
import static io.deephaven.util.type.TypeUtils.unbox;
@@ -40,9 +41,11 @@ public byte getByte(final long rowKey) {
interface MakeRegionDefault extends MakeRegion> {
@Override
- default ColumnRegionByte makeRegion(@NotNull final ColumnDefinition> columnDefinition,
- @NotNull final ColumnLocation columnLocation,
- final int regionIndex) {
+ default ColumnRegionByte makeRegion(
+ @NotNull final ColumnDefinition> columnDefinition,
+ @NotNull final ColumnLocation columnLocation,
+ @NotNull final SourceTableColumnInstructions instructions,
+ final int regionIndex) {
if (columnLocation.exists()) {
return columnLocation.makeColumnRegionByte(columnDefinition);
}
@@ -78,9 +81,11 @@ static final class Partitioning extends RegionedColumnSourceByte {
}
@Override
- public ColumnRegionByte makeRegion(@NotNull final ColumnDefinition> columnDefinition,
- @NotNull final ColumnLocation columnLocation,
- final int regionIndex) {
+ public ColumnRegionByte makeRegion(
+ @NotNull final ColumnDefinition> columnDefinition,
+ @NotNull final ColumnLocation columnLocation,
+ @NotNull final SourceTableColumnInstructions instructions,
+ final int regionIndex) {
final TableLocationKey locationKey = columnLocation.getTableLocation().getKey();
final Object partitioningColumnValue = locationKey.getPartitionValue(columnDefinition.getName());
if (partitioningColumnValue != null && !Byte.class.isAssignableFrom(partitioningColumnValue.getClass())) {
diff --git a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceChar.java b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceChar.java
index 0316daefc43..e16bb491d74 100644
--- a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceChar.java
+++ b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceChar.java
@@ -10,6 +10,7 @@
import io.deephaven.engine.table.impl.locations.TableLocationKey;
import io.deephaven.engine.table.impl.ColumnSourceGetDefaults;
import io.deephaven.chunk.attributes.Values;
+import io.deephaven.engine.table.impl.sources.regioned.instructions.SourceTableColumnInstructions;
import org.jetbrains.annotations.NotNull;
import static io.deephaven.util.type.TypeUtils.unbox;
@@ -33,9 +34,11 @@ public char getChar(final long rowKey) {
interface MakeRegionDefault extends MakeRegion> {
@Override
- default ColumnRegionChar makeRegion(@NotNull final ColumnDefinition> columnDefinition,
- @NotNull final ColumnLocation columnLocation,
- final int regionIndex) {
+ default ColumnRegionChar makeRegion(
+ @NotNull final ColumnDefinition> columnDefinition,
+ @NotNull final ColumnLocation columnLocation,
+ @NotNull final SourceTableColumnInstructions instructions,
+ final int regionIndex) {
if (columnLocation.exists()) {
return columnLocation.makeColumnRegionChar(columnDefinition);
}
@@ -61,9 +64,11 @@ static final class Partitioning extends RegionedColumnSourceChar {
}
@Override
- public ColumnRegionChar makeRegion(@NotNull final ColumnDefinition> columnDefinition,
- @NotNull final ColumnLocation columnLocation,
- final int regionIndex) {
+ public ColumnRegionChar makeRegion(
+ @NotNull final ColumnDefinition> columnDefinition,
+ @NotNull final ColumnLocation columnLocation,
+ @NotNull final SourceTableColumnInstructions instructions,
+ final int regionIndex) {
final TableLocationKey locationKey = columnLocation.getTableLocation().getKey();
final Object partitioningColumnValue = locationKey.getPartitionValue(columnDefinition.getName());
if (partitioningColumnValue != null && !Character.class.isAssignableFrom(partitioningColumnValue.getClass())) {
diff --git a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceDouble.java b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceDouble.java
index 6c50a7cc3a0..6d42301e982 100644
--- a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceDouble.java
+++ b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceDouble.java
@@ -15,6 +15,7 @@
import io.deephaven.engine.table.impl.locations.TableLocationKey;
import io.deephaven.engine.table.impl.ColumnSourceGetDefaults;
import io.deephaven.chunk.attributes.Values;
+import io.deephaven.engine.table.impl.sources.regioned.instructions.SourceTableColumnInstructions;
import org.jetbrains.annotations.NotNull;
import static io.deephaven.util.type.TypeUtils.unbox;
@@ -38,9 +39,11 @@ public double getDouble(final long rowKey) {
interface MakeRegionDefault extends MakeRegion> {
@Override
- default ColumnRegionDouble makeRegion(@NotNull final ColumnDefinition> columnDefinition,
- @NotNull final ColumnLocation columnLocation,
- final int regionIndex) {
+ default ColumnRegionDouble makeRegion(
+ @NotNull final ColumnDefinition> columnDefinition,
+ @NotNull final ColumnLocation columnLocation,
+ @NotNull final SourceTableColumnInstructions instructions,
+ final int regionIndex) {
if (columnLocation.exists()) {
return columnLocation.makeColumnRegionDouble(columnDefinition);
}
@@ -66,9 +69,11 @@ static final class Partitioning extends RegionedColumnSourceDouble {
}
@Override
- public ColumnRegionDouble makeRegion(@NotNull final ColumnDefinition> columnDefinition,
- @NotNull final ColumnLocation columnLocation,
- final int regionIndex) {
+ public ColumnRegionDouble makeRegion(
+ @NotNull final ColumnDefinition> columnDefinition,
+ @NotNull final ColumnLocation columnLocation,
+ @NotNull final SourceTableColumnInstructions instructions,
+ final int regionIndex) {
final TableLocationKey locationKey = columnLocation.getTableLocation().getKey();
final Object partitioningColumnValue = locationKey.getPartitionValue(columnDefinition.getName());
if (partitioningColumnValue != null && !Double.class.isAssignableFrom(partitioningColumnValue.getClass())) {
diff --git a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceFloat.java b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceFloat.java
index ae8bb9184dc..d9ce04968c2 100644
--- a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceFloat.java
+++ b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceFloat.java
@@ -15,6 +15,7 @@
import io.deephaven.engine.table.impl.locations.TableLocationKey;
import io.deephaven.engine.table.impl.ColumnSourceGetDefaults;
import io.deephaven.chunk.attributes.Values;
+import io.deephaven.engine.table.impl.sources.regioned.instructions.SourceTableColumnInstructions;
import org.jetbrains.annotations.NotNull;
import static io.deephaven.util.type.TypeUtils.unbox;
@@ -38,9 +39,11 @@ public float getFloat(final long rowKey) {
interface MakeRegionDefault extends MakeRegion> {
@Override
- default ColumnRegionFloat makeRegion(@NotNull final ColumnDefinition> columnDefinition,
- @NotNull final ColumnLocation columnLocation,
- final int regionIndex) {
+ default ColumnRegionFloat makeRegion(
+ @NotNull final ColumnDefinition> columnDefinition,
+ @NotNull final ColumnLocation columnLocation,
+ @NotNull final SourceTableColumnInstructions instructions,
+ final int regionIndex) {
if (columnLocation.exists()) {
return columnLocation.makeColumnRegionFloat(columnDefinition);
}
@@ -66,9 +69,11 @@ static final class Partitioning extends RegionedColumnSourceFloat {
}
@Override
- public ColumnRegionFloat makeRegion(@NotNull final ColumnDefinition> columnDefinition,
- @NotNull final ColumnLocation columnLocation,
- final int regionIndex) {
+ public ColumnRegionFloat makeRegion(
+ @NotNull final ColumnDefinition> columnDefinition,
+ @NotNull final ColumnLocation columnLocation,
+ @NotNull final SourceTableColumnInstructions instructions,
+ final int regionIndex) {
final TableLocationKey locationKey = columnLocation.getTableLocation().getKey();
final Object partitioningColumnValue = locationKey.getPartitionValue(columnDefinition.getName());
if (partitioningColumnValue != null && !Float.class.isAssignableFrom(partitioningColumnValue.getClass())) {
diff --git a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceInstant.java b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceInstant.java
index d90c58bb209..d0126fd41f9 100644
--- a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceInstant.java
+++ b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceInstant.java
@@ -87,4 +87,16 @@ public ColumnSource toLocalTime(ZoneId zone) {
public ColumnSource toEpochNano() {
return getNativeSource();
}
+
+ @Override
+ @NotNull
+ public Object[] convertArray(@NotNull final Object[] keys) {
+ final Object[] newValues = new Object[keys.length];
+ for (int ii = 0; ii < keys.length; ii++) {
+ if (keys[ii] == null || keys[ii] instanceof Instant) {
+ newValues[ii] = DateTimeUtils.epochNanos((Instant) keys[ii]);
+ }
+ }
+ return newValues;
+ }
}
diff --git a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceInt.java b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceInt.java
index f07e2d05733..d11255995e5 100644
--- a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceInt.java
+++ b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceInt.java
@@ -15,6 +15,7 @@
import io.deephaven.engine.table.impl.locations.TableLocationKey;
import io.deephaven.engine.table.impl.ColumnSourceGetDefaults;
import io.deephaven.chunk.attributes.Values;
+import io.deephaven.engine.table.impl.sources.regioned.instructions.SourceTableColumnInstructions;
import org.jetbrains.annotations.NotNull;
import static io.deephaven.util.type.TypeUtils.unbox;
@@ -38,9 +39,11 @@ public int getInt(final long rowKey) {
interface MakeRegionDefault extends MakeRegion> {
@Override
- default ColumnRegionInt makeRegion(@NotNull final ColumnDefinition> columnDefinition,
- @NotNull final ColumnLocation columnLocation,
- final int regionIndex) {
+ default ColumnRegionInt makeRegion(
+ @NotNull final ColumnDefinition> columnDefinition,
+ @NotNull final ColumnLocation columnLocation,
+ @NotNull final SourceTableColumnInstructions instructions,
+ final int regionIndex) {
if (columnLocation.exists()) {
return columnLocation.makeColumnRegionInt(columnDefinition);
}
@@ -66,9 +69,11 @@ static final class Partitioning extends RegionedColumnSourceInt {
}
@Override
- public ColumnRegionInt makeRegion(@NotNull final ColumnDefinition> columnDefinition,
- @NotNull final ColumnLocation columnLocation,
- final int regionIndex) {
+ public ColumnRegionInt makeRegion(
+ @NotNull final ColumnDefinition> columnDefinition,
+ @NotNull final ColumnLocation columnLocation,
+ @NotNull final SourceTableColumnInstructions instructions,
+ final int regionIndex) {
final TableLocationKey locationKey = columnLocation.getTableLocation().getKey();
final Object partitioningColumnValue = locationKey.getPartitionValue(columnDefinition.getName());
if (partitioningColumnValue != null && !Integer.class.isAssignableFrom(partitioningColumnValue.getClass())) {
diff --git a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceLong.java b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceLong.java
index 9009869c1bb..619753914bb 100644
--- a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceLong.java
+++ b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceLong.java
@@ -26,6 +26,7 @@
import io.deephaven.engine.table.impl.locations.TableLocationKey;
import io.deephaven.engine.table.impl.ColumnSourceGetDefaults;
import io.deephaven.chunk.attributes.Values;
+import io.deephaven.engine.table.impl.sources.regioned.instructions.SourceTableColumnInstructions;
import org.jetbrains.annotations.NotNull;
import static io.deephaven.util.type.TypeUtils.unbox;
@@ -49,9 +50,11 @@ public long getLong(final long rowKey) {
interface MakeRegionDefault extends MakeRegion> {
@Override
- default ColumnRegionLong makeRegion(@NotNull final ColumnDefinition> columnDefinition,
- @NotNull final ColumnLocation columnLocation,
- final int regionIndex) {
+ default ColumnRegionLong makeRegion(
+ @NotNull final ColumnDefinition> columnDefinition,
+ @NotNull final ColumnLocation columnLocation,
+ @NotNull final SourceTableColumnInstructions instructions,
+ final int regionIndex) {
if (columnLocation.exists()) {
return columnLocation.makeColumnRegionLong(columnDefinition);
}
@@ -126,9 +129,11 @@ static final class Partitioning extends RegionedColumnSourceLong {
}
@Override
- public ColumnRegionLong makeRegion(@NotNull final ColumnDefinition> columnDefinition,
- @NotNull final ColumnLocation columnLocation,
- final int regionIndex) {
+ public ColumnRegionLong makeRegion(
+ @NotNull final ColumnDefinition> columnDefinition,
+ @NotNull final ColumnLocation columnLocation,
+ @NotNull final SourceTableColumnInstructions instructions,
+ final int regionIndex) {
final TableLocationKey locationKey = columnLocation.getTableLocation().getKey();
final Object partitioningColumnValue = locationKey.getPartitionValue(columnDefinition.getName());
if (partitioningColumnValue != null && !Long.class.isAssignableFrom(partitioningColumnValue.getClass())) {
diff --git a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceManager.java b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceManager.java
index 572e36a0313..0812cfc8fb3 100644
--- a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceManager.java
+++ b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceManager.java
@@ -8,16 +8,18 @@
import io.deephaven.engine.rowset.RowSetBuilderSequential;
import io.deephaven.engine.rowset.RowSetFactory;
import io.deephaven.engine.rowset.WritableRowSet;
-import io.deephaven.engine.table.ColumnDefinition;
-import io.deephaven.engine.table.impl.ColumnSourceManager;
import io.deephaven.engine.table.impl.ColumnToCodecMappings;
-import io.deephaven.engine.table.impl.locations.*;
import io.deephaven.engine.table.impl.locations.impl.TableLocationUpdateSubscriptionBuffer;
-import io.deephaven.engine.table.impl.sources.DeferredGroupingColumnSource;
+import io.deephaven.engine.table.impl.sources.regioned.instructions.SourceTableColumnInstructions;
+import io.deephaven.engine.table.impl.sources.regioned.instructions.SourceTableInstructions;
import io.deephaven.hash.KeyedObjectHashMap;
import io.deephaven.hash.KeyedObjectKey;
-import io.deephaven.internal.log.LoggerFactory;
import io.deephaven.io.logger.Logger;
+import io.deephaven.engine.table.ColumnDefinition;
+import io.deephaven.engine.table.impl.ColumnSourceManager;
+import io.deephaven.engine.table.impl.locations.*;
+import io.deephaven.engine.table.impl.sources.DeferredGroupingColumnSource;
+import io.deephaven.internal.log.LoggerFactory;
import org.jetbrains.annotations.NotNull;
import java.util.*;
@@ -49,7 +51,7 @@ public class RegionedColumnSourceManager implements ColumnSourceManager {
/**
* An unmodifiable view of columnSources.
*/
- private final Map> sharedColumnSources =
+ private final Map> sharedColumnSources =
Collections.unmodifiableMap(columnSources);
/**
@@ -69,6 +71,9 @@ public class RegionedColumnSourceManager implements ColumnSourceManager {
*/
private final List orderedIncludedTableLocations = new ArrayList<>();
+ /** A control class for modifying the behavior of the table and its column sources */
+ private final SourceTableInstructions instructions;
+
/**
* Whether grouping is enabled.
*/
@@ -82,16 +87,38 @@ public class RegionedColumnSourceManager implements ColumnSourceManager {
* @param componentFactory The component factory
* @param columnDefinitions The column definitions
*/
- RegionedColumnSourceManager(final boolean isRefreshing,
+ RegionedColumnSourceManager(
+ final boolean isRefreshing,
+ @NotNull final RegionedTableComponentFactory componentFactory,
+ @NotNull final ColumnToCodecMappings codecMappings,
+ @NotNull final List> columnDefinitions) {
+ this(isRefreshing, componentFactory, codecMappings, SourceTableInstructions.EMPTY, columnDefinitions);
+ }
+
+ /**
+ * Construct a column manager with the specified component factory and definitions.
+ *
+ * @param isRefreshing Whether the table using this column source manager is refreshing
+ * @param componentFactory The component factory
+ * @param instructions A control class for modifying the behavior of the table and its column sources
+ * @param columnDefinitions The column definitions
+ */
+ RegionedColumnSourceManager(
+ final boolean isRefreshing,
@NotNull final RegionedTableComponentFactory componentFactory,
@NotNull final ColumnToCodecMappings codecMappings,
+ @NotNull final SourceTableInstructions instructions,
@NotNull final List> columnDefinitions) {
this.isRefreshing = isRefreshing;
+ this.instructions = instructions;
this.columnDefinitions = columnDefinitions;
for (final ColumnDefinition> columnDefinition : columnDefinitions) {
- columnSources.put(
- columnDefinition.getName(),
- componentFactory.createRegionedColumnSource(columnDefinition, codecMappings));
+ final RegionedColumnSource> regionedColumnSource =
+ componentFactory.createRegionedColumnSource(columnDefinition, codecMappings);
+ if (instructions.groupingDisabled()) {
+ regionedColumnSource.disableGrouping();
+ }
+ columnSources.put(columnDefinition.getName(), regionedColumnSource);
}
}
@@ -199,7 +226,7 @@ public final synchronized boolean isEmpty() {
}
@Override
- public final Map> getColumnSources() {
+ public final Map> getColumnSources() {
return sharedColumnSources;
}
@@ -211,7 +238,8 @@ public final synchronized void disableGrouping() {
isGroupingEnabled = false;
for (ColumnDefinition> columnDefinition : columnDefinitions) {
if (columnDefinition.isGrouping()) {
- DeferredGroupingColumnSource> columnSource = getColumnSources().get(columnDefinition.getName());
+ final RegionedColumnSource> columnSource = getColumnSources().get(columnDefinition.getName());
+ columnSource.disableGrouping();
columnSource.setGroupingProvider(null);
columnSource.setGroupToRange(null);
}
@@ -304,12 +332,13 @@ private void processInitial(final RowSetBuilderSequential addedRowSetBuilder, fi
.appendRange(regionFirstKey + subRegionFirstKey, regionFirstKey + subRegionLastKey));
RowSet addRowSetInTable = null;
try {
- for (final ColumnDefinition columnDefinition : columnDefinitions) {
- // noinspection unchecked
+ for (final ColumnDefinition> columnDefinition : columnDefinitions) {
+ // noinspection unchecked,rawtypes
final ColumnLocationState state = new ColumnLocationState(
columnDefinition,
columnSources.get(columnDefinition.getName()),
- location.getColumnLocation(columnDefinition.getName()));
+ location.getColumnLocation(columnDefinition.getName()),
+ instructions.getInstructions(columnDefinition.getName()));
columnLocationStates.add(state);
state.regionAllocated(regionIndex);
if (state.needToUpdateGrouping()) {
@@ -430,18 +459,23 @@ private class ColumnLocationState {
protected final ColumnDefinition definition;
protected final RegionedColumnSource source;
protected final ColumnLocation location;
-
- private ColumnLocationState(ColumnDefinition definition,
- RegionedColumnSource source,
- ColumnLocation location) {
+ @NotNull
+ private final SourceTableColumnInstructions instructions;
+
+ private ColumnLocationState(
+ @NotNull final ColumnDefinition definition,
+ @NotNull final RegionedColumnSource source,
+ @NotNull final ColumnLocation location,
+ @NotNull final SourceTableColumnInstructions instructions) {
this.definition = definition;
this.source = source;
this.location = location;
+ this.instructions = instructions;
}
private void regionAllocated(final int regionIndex) {
- Assert.eq(regionIndex, "regionIndex", source.addRegion(definition, location),
- "source.addRegion((definition, location)");
+ Assert.eq(regionIndex, "regionIndex", source.addRegion(definition, location, instructions),
+ "source.addRegion((definition, location, instructions)");
}
private boolean needToUpdateGrouping() {
@@ -456,6 +490,7 @@ private boolean needToUpdateGrouping() {
private void updateGrouping(@NotNull final RowSet locationAddedRowSetInTable) {
if (definition.isGrouping()) {
Assert.eqTrue(isGroupingEnabled, "isGroupingEnabled");
+ // noinspection rawtypes
GroupingProvider groupingProvider = source.getGroupingProvider();
if (groupingProvider == null) {
groupingProvider = GroupingProvider.makeGroupingProvider(definition);
@@ -463,7 +498,7 @@ private void updateGrouping(@NotNull final RowSet locationAddedRowSetInTable) {
source.setGroupingProvider(groupingProvider);
}
if (groupingProvider instanceof KeyRangeGroupingProvider) {
- ((KeyRangeGroupingProvider) groupingProvider).addSource(location, locationAddedRowSetInTable);
+ ((KeyRangeGroupingProvider>) groupingProvider).addSource(location, locationAddedRowSetInTable);
}
} else if (definition.isPartitioning()) {
final DeferredGroupingColumnSource partitioningColumnSource = source;
diff --git a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceObject.java b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceObject.java
index b73e5bdf8a6..e5a1e34b40f 100644
--- a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceObject.java
+++ b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceObject.java
@@ -10,6 +10,7 @@
import io.deephaven.engine.table.impl.locations.TableLocationKey;
import io.deephaven.engine.table.impl.ColumnSourceGetDefaults;
import io.deephaven.chunk.attributes.Values;
+import io.deephaven.engine.table.impl.sources.regioned.instructions.SourceTableColumnInstructions;
import org.jetbrains.annotations.NotNull;
import org.jetbrains.annotations.Nullable;
@@ -39,9 +40,12 @@ public AsValues(@NotNull final Class dataType, @Nullable final Class<
super(ColumnRegionObject.createNull(PARAMETERS.regionMask), dataType, componentType, DeferredColumnRegionObject::new);
}
- public ColumnRegionObject makeRegion(@NotNull final ColumnDefinition> columnDefinition,
- @NotNull final ColumnLocation columnLocation,
- final int regionIndex) {
+ public ColumnRegionObject makeRegion(
+ @NotNull final ColumnDefinition> columnDefinition,
+ @NotNull final ColumnLocation columnLocation,
+ @NotNull final SourceTableColumnInstructions instructions,
+ final int regionIndex) {
+ // TODO NATE NOCOMMIT: propagate instructions to region creation
if (columnLocation.exists()) {
//noinspection unchecked
return (ColumnRegionObject) columnLocation.makeColumnRegionObject(columnDefinition);
@@ -59,9 +63,11 @@ static final class Partitioning extends RegionedColumnSourceObject makeRegion(@NotNull final ColumnDefinition> columnDefinition,
- @NotNull final ColumnLocation columnLocation,
- final int regionIndex) {
+ public ColumnRegionObject makeRegion(
+ @NotNull final ColumnDefinition> columnDefinition,
+ @NotNull final ColumnLocation columnLocation,
+ @NotNull final SourceTableColumnInstructions instructions,
+ final int regionIndex) {
final TableLocationKey locationKey = columnLocation.getTableLocation().getKey();
final Object partitioningColumnValue = locationKey.getPartitionValue(columnDefinition.getName());
if (partitioningColumnValue != null && !getType().isAssignableFrom(partitioningColumnValue.getClass())) {
diff --git a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceReferencing.java b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceReferencing.java
index b963c238557..80059d626d4 100644
--- a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceReferencing.java
+++ b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceReferencing.java
@@ -8,7 +8,9 @@
import io.deephaven.engine.table.impl.locations.ColumnLocation;
import io.deephaven.engine.table.ColumnSource;
import io.deephaven.engine.table.SharedContext;
+import io.deephaven.engine.table.impl.sources.regioned.instructions.SourceTableColumnInstructions;
import org.jetbrains.annotations.NotNull;
+import org.jetbrains.annotations.Nullable;
import javax.annotation.OverridingMethodsMustInvokeSuper;
@@ -18,7 +20,7 @@
*/
abstract class RegionedColumnSourceReferencing>
extends RegionedColumnSourceBase>
- implements ColumnRegionReferencingImpl.Converter {
+ implements ColumnRegionReferencing.Converter {
@NotNull
private final ColumnRegionReferencing.Null nullRegion;
@@ -57,7 +59,7 @@ public ColumnRegionReferencing getRegion(int regionInd
return nullRegion;
}
- return new ColumnRegionReferencingImpl<>(nativeRegion);
+ return new ColumnRegionReferencingImpl<>(nativeRegion, this);
}
@Override
@@ -66,8 +68,11 @@ public int getRegionCount() {
}
@Override
- public int addRegion(@NotNull ColumnDefinition> columnDefinition, @NotNull ColumnLocation columnLocation) {
- return nativeSource.addRegion(columnDefinition, columnLocation);
+ public int addRegion(
+ @NotNull final ColumnDefinition> columnDefinition,
+ @NotNull final ColumnLocation columnLocation,
+ @NotNull final SourceTableColumnInstructions instructions) {
+ return nativeSource.addRegion(columnDefinition, columnLocation, instructions);
}
@Override
@@ -76,8 +81,10 @@ int addRegionForUnitTests(OTHER_REGION_TYPE region) {
}
@Override
- public FillContext makeFillContext(int chunkCapacity, SharedContext sharedContext) {
- return new ColumnRegionReferencingImpl.FillContext<>(nativeSource, this, chunkCapacity, sharedContext);
+ public FillContext makeFillContext(
+ final int chunkCapacity,
+ @Nullable final SharedContext sharedContext) {
+ return new ColumnRegionReferencingImpl.FillContext<>(nativeSource, chunkCapacity, sharedContext);
}
@NotNull
diff --git a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceShort.java b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceShort.java
index 181984c3ea3..78b7446eea8 100644
--- a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceShort.java
+++ b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceShort.java
@@ -15,6 +15,7 @@
import io.deephaven.engine.table.impl.locations.TableLocationKey;
import io.deephaven.engine.table.impl.ColumnSourceGetDefaults;
import io.deephaven.chunk.attributes.Values;
+import io.deephaven.engine.table.impl.sources.regioned.instructions.SourceTableColumnInstructions;
import org.jetbrains.annotations.NotNull;
import static io.deephaven.util.type.TypeUtils.unbox;
@@ -38,9 +39,11 @@ public short getShort(final long rowKey) {
interface MakeRegionDefault extends MakeRegion> {
@Override
- default ColumnRegionShort makeRegion(@NotNull final ColumnDefinition> columnDefinition,
- @NotNull final ColumnLocation columnLocation,
- final int regionIndex) {
+ default ColumnRegionShort makeRegion(
+ @NotNull final ColumnDefinition> columnDefinition,
+ @NotNull final ColumnLocation columnLocation,
+ @NotNull final SourceTableColumnInstructions instructions,
+ final int regionIndex) {
if (columnLocation.exists()) {
return columnLocation.makeColumnRegionShort(columnDefinition);
}
@@ -66,9 +69,11 @@ static final class Partitioning extends RegionedColumnSourceShort {
}
@Override
- public ColumnRegionShort makeRegion(@NotNull final ColumnDefinition> columnDefinition,
- @NotNull final ColumnLocation columnLocation,
- final int regionIndex) {
+ public ColumnRegionShort makeRegion(
+ @NotNull final ColumnDefinition> columnDefinition,
+ @NotNull final ColumnLocation columnLocation,
+ @NotNull final SourceTableColumnInstructions instructions,
+ final int regionIndex) {
final TableLocationKey locationKey = columnLocation.getTableLocation().getKey();
final Object partitioningColumnValue = locationKey.getPartitionValue(columnDefinition.getName());
if (partitioningColumnValue != null && !Short.class.isAssignableFrom(partitioningColumnValue.getClass())) {
diff --git a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceWithDictionary.java b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceWithDictionary.java
index c9bd05ddbee..5072313ef75 100644
--- a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceWithDictionary.java
+++ b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceWithDictionary.java
@@ -15,6 +15,7 @@
import io.deephaven.engine.table.impl.sources.RowIdSource;
import io.deephaven.engine.table.impl.chunkattributes.DictionaryKeys;
import io.deephaven.chunk.attributes.Values;
+import io.deephaven.engine.table.impl.sources.regioned.instructions.SourceTableColumnInstructions;
import org.apache.commons.lang3.mutable.Mutable;
import org.apache.commons.lang3.mutable.MutableObject;
import org.jetbrains.annotations.NotNull;
@@ -61,12 +62,10 @@ private final class AsLong
extends RegionedColumnSourceBase>
implements ColumnSourceGetDefaults.ForLong {
- private final ColumnRegionLong nullRegion;
private volatile ColumnRegionLong[] wrapperRegions;
private AsLong() {
super(long.class);
- nullRegion = ColumnRegionLong.createNull(PARAMETERS.regionMask);
// noinspection unchecked
wrapperRegions = new ColumnRegionLong[0];
}
@@ -78,9 +77,11 @@ public long getLong(final long rowKey) {
}
@Override
- public int addRegion(@NotNull final ColumnDefinition> columnDefinition,
- @NotNull final ColumnLocation columnLocation) {
- return RegionedColumnSourceWithDictionary.this.addRegion(columnDefinition, columnLocation);
+ public int addRegion(
+ @NotNull final ColumnDefinition> columnDefinition,
+ @NotNull final ColumnLocation columnLocation,
+ @NotNull final SourceTableColumnInstructions instructions) {
+ return RegionedColumnSourceWithDictionary.this.addRegion(columnDefinition, columnLocation, instructions);
}
@Override
@@ -91,7 +92,7 @@ int addRegionForUnitTests(@NotNull final OTHER_REGION_TYPE r
@NotNull
@Override
ColumnRegionLong getNullRegion() {
- return nullRegion;
+ return ColumnRegionLong.createNull(PARAMETERS.regionMask);
}
@Override
@@ -104,7 +105,7 @@ public ColumnRegionLong getRegion(final int regionIndex) {
final ColumnRegionObject sourceRegion =
RegionedColumnSourceWithDictionary.this.getRegion(regionIndex);
if (sourceRegion instanceof ColumnRegion.Null) {
- return nullRegion;
+ return getNullRegion();
}
ColumnRegionLong[] localWrappers;
ColumnRegionLong wrapper;
@@ -168,9 +169,11 @@ public DATA_TYPE get(final long rowKey) {
}
@Override
- public int addRegion(@NotNull final ColumnDefinition> columnDefinition,
- @NotNull final ColumnLocation columnLocation) {
- return RegionedColumnSourceWithDictionary.this.addRegion(columnDefinition, columnLocation);
+ public int addRegion(
+ @NotNull final ColumnDefinition> columnDefinition,
+ @NotNull final ColumnLocation columnLocation,
+ @NotNull final SourceTableColumnInstructions instructions) {
+ return RegionedColumnSourceWithDictionary.this.addRegion(columnDefinition, columnLocation, instructions);
}
@Override
diff --git a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceZonedDateTime.java b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceZonedDateTime.java
index 333366aaee5..3484b2c5092 100644
--- a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceZonedDateTime.java
+++ b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedColumnSourceZonedDateTime.java
@@ -105,4 +105,16 @@ public ColumnSource toEpochNano() {
public ZoneId getZone() {
return zone;
}
+
+ @Override
+ @NotNull
+ public Object[] convertArray(@NotNull final Object[] keys) {
+ final Object[] convertedValues = new Object[keys.length];
+ for (int ii = 0; ii < keys.length; ii++) {
+ if (keys[ii] == null || keys[ii] instanceof ZonedDateTime) {
+ convertedValues[ii] = DateTimeUtils.epochNanos((ZonedDateTime) keys[ii]);
+ }
+ }
+ return convertedValues;
+ }
}
diff --git a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedPageStore.java b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedPageStore.java
index b429158ba07..22b201fe45b 100644
--- a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedPageStore.java
+++ b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedPageStore.java
@@ -10,9 +10,12 @@
import io.deephaven.engine.page.Page;
import io.deephaven.engine.page.PageStore;
import io.deephaven.engine.rowset.RowSequence;
+import io.deephaven.engine.table.impl.locations.ColumnLocation;
import io.deephaven.util.annotations.FinalDefault;
import org.jetbrains.annotations.NotNull;
+import java.util.Arrays;
+
public interface RegionedPageStore>
extends PageStore {
@@ -133,18 +136,22 @@ private static long validateMask(final long mask, final String name) {
/**
* A regioned page store for use when the full set of regions and their sizes are known.
*/
- abstract class Static>
+ abstract class Static>
implements RegionedPageStore {
private final Parameters parameters;
private final REGION_TYPE[] regions;
+ @NotNull
+ private final ColumnLocation columnLocation;
/**
* @param parameters Mask and shift parameters
* @param regions Array of all regions in this page store. Array becomes property of the page store.
*/
- public Static(@NotNull final Parameters parameters,
- @NotNull final REGION_TYPE[] regions) {
+ public Static(
+ @NotNull final Parameters parameters,
+ @NotNull final REGION_TYPE[] regions,
+ @NotNull final ColumnLocation columnLocation) {
this.parameters = parameters;
this.regions = Require.elementsNeqNull(regions, "regions");
Require.leq(regions.length, "regions.length", parameters.maximumRegionCount,
@@ -152,6 +159,7 @@ public Static(@NotNull final Parameters parameters,
for (final REGION_TYPE region : regions) {
Require.eq(region.mask(), "region.mask()", parameters.regionMask, "parameters.regionMask");
}
+ this.columnLocation = columnLocation;
}
@Override
@@ -168,5 +176,9 @@ public final int getRegionCount() {
public final REGION_TYPE getRegion(final int regionIndex) {
return regions[regionIndex];
}
+
+ public ColumnLocation getLocation() {
+ return columnLocation;
+ }
}
}
diff --git a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedTableComponentFactoryImpl.java b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedTableComponentFactoryImpl.java
index 70a7154f799..106598722de 100644
--- a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedTableComponentFactoryImpl.java
+++ b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/RegionedTableComponentFactoryImpl.java
@@ -7,6 +7,7 @@
import io.deephaven.engine.table.impl.locations.TableDataException;
import io.deephaven.engine.table.impl.ColumnSourceManager;
import io.deephaven.engine.table.impl.ColumnToCodecMappings;
+import io.deephaven.engine.table.impl.sources.regioned.instructions.SourceTableInstructions;
import io.deephaven.util.type.TypeUtils;
import org.jetbrains.annotations.NotNull;
@@ -24,6 +25,11 @@ public class RegionedTableComponentFactoryImpl implements RegionedTableComponent
private static final Map, Supplier>> SIMPLE_DATA_TYPE_TO_REGIONED_COLUMN_SOURCE_SUPPLIER;
+ private static final RegionedTableComponentFactory INSTANCE =
+ new RegionedTableComponentFactoryImpl(SourceTableInstructions.EMPTY);
+
+ private final SourceTableInstructions instructions;
+
static {
Map, Supplier>> typeToSupplier = new HashMap<>();
typeToSupplier.put(Byte.class, RegionedColumnSourceByte.AsValues::new);
@@ -38,16 +44,35 @@ public class RegionedTableComponentFactoryImpl implements RegionedTableComponent
SIMPLE_DATA_TYPE_TO_REGIONED_COLUMN_SOURCE_SUPPLIER = Collections.unmodifiableMap(typeToSupplier);
}
- public static final RegionedTableComponentFactory INSTANCE = new RegionedTableComponentFactoryImpl();
+ /**
+ * Create a default instance of the factory, using no special table or column instructions.
+ *
+ * @return a {RegionedTableComponentFactory}
+ */
+ public static RegionedTableComponentFactory make() {
+ return INSTANCE;
+ }
- private RegionedTableComponentFactoryImpl() {}
+ /**
+ * Create a factory that uses the input {@link SourceTableInstructions} for column and region creation.
+ *
+ * @param instructions the instructions
+ * @return a new {@link RegionedTableComponentFactory}
+ */
+ public static RegionedTableComponentFactory make(@NotNull final SourceTableInstructions instructions) {
+ return new RegionedTableComponentFactoryImpl(instructions);
+ }
+
+ private RegionedTableComponentFactoryImpl(@NotNull final SourceTableInstructions instructions) {
+ this.instructions = instructions;
+ }
@Override
public ColumnSourceManager createColumnSourceManager(
final boolean isRefreshing,
@NotNull final ColumnToCodecMappings codecMappings,
@NotNull final List> columnDefinitions) {
- return new RegionedColumnSourceManager(isRefreshing, this, codecMappings, columnDefinitions);
+ return new RegionedColumnSourceManager(isRefreshing, this, codecMappings, instructions, columnDefinitions);
}
/**
diff --git a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/instructions/CacheBehavior.java b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/instructions/CacheBehavior.java
new file mode 100644
index 00000000000..25986a4fd81
--- /dev/null
+++ b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/instructions/CacheBehavior.java
@@ -0,0 +1,15 @@
+/**
+ * Copyright (c) 2016-2023 Deephaven Data Labs and Patent Pending
+ */
+package io.deephaven.engine.table.impl.sources.regioned.instructions;
+
+public enum CacheBehavior {
+ /** Use Deephaven's normal caching behavior */
+ Default,
+
+ /** Always cache results */
+ Always,
+
+ /** Never cache results */
+ Never
+}
diff --git a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/instructions/CacheType.java b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/instructions/CacheType.java
new file mode 100644
index 00000000000..64abdcb0fb9
--- /dev/null
+++ b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/instructions/CacheType.java
@@ -0,0 +1,13 @@
+/**
+ * Copyright (c) 2016-2023 Deephaven Data Labs and Patent Pending
+ */
+package io.deephaven.engine.table.impl.sources.regioned.instructions;
+
+public enum CacheType {
+
+ /** Create caches using soft references */
+ Soft,
+
+ /** Create caches using hard references */
+ Hard
+}
diff --git a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/instructions/SourceTableColumnInstructions.java b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/instructions/SourceTableColumnInstructions.java
new file mode 100644
index 00000000000..def0a10b8e6
--- /dev/null
+++ b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/instructions/SourceTableColumnInstructions.java
@@ -0,0 +1,40 @@
+/**
+ * Copyright (c) 2016-2023 Deephaven Data Labs and Patent Pending
+ */
+package io.deephaven.engine.table.impl.sources.regioned.instructions;
+
+import io.deephaven.annotations.BuildableStyle;
+import org.immutables.value.Value;
+import org.jetbrains.annotations.NotNull;
+
+
+@Value.Immutable
+@BuildableStyle
+public abstract class SourceTableColumnInstructions {
+ public static final SourceTableColumnInstructions DEFAULT = builder().build();
+
+ /**
+ * @return the desired caching behavior
+ */
+ @Value.Default
+ public CacheBehavior cacheBehavior() {
+ return CacheBehavior.Default;
+ }
+
+ @Value.Default
+ public CacheType cacheType() {
+ return CacheType.Soft;
+ }
+
+ public static Builder builder() {
+ return ImmutableSourceTableColumnInstructions.builder();
+ }
+
+ public interface Builder {
+ Builder cacheBehavior(@NotNull CacheBehavior cacheBehavior);
+
+ Builder cacheType(@NotNull CacheType cacheType);
+
+ SourceTableColumnInstructions build();
+ }
+}
diff --git a/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/instructions/SourceTableInstructions.java b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/instructions/SourceTableInstructions.java
new file mode 100644
index 00000000000..1ccea67588e
--- /dev/null
+++ b/engine/table/src/main/java/io/deephaven/engine/table/impl/sources/regioned/instructions/SourceTableInstructions.java
@@ -0,0 +1,44 @@
+package io.deephaven.engine.table.impl.sources.regioned.instructions;
+
+import io.deephaven.annotations.BuildableStyle;
+import io.deephaven.engine.table.impl.SourceTable;
+import org.immutables.value.Value;
+import org.jetbrains.annotations.NotNull;
+
+import java.util.Map;
+
+/**
+ * A contextual class to control the underlying behaviors of {@link SourceTable source tables}. This can be used to
+ * control the behaviors of different region types, for example Parquet and Deephaven regions. It can also be used to
+ * control behaviors of the source table itself.
+ */
+
+@Value.Immutable
+@BuildableStyle
+public abstract class SourceTableInstructions {
+ public static final SourceTableInstructions EMPTY = builder().build();
+
+ @Value.Default
+ public boolean groupingDisabled() {
+ return false;
+ }
+
+ abstract Map columnInstructions();
+
+ public static Builder builder() {
+ return ImmutableSourceTableInstructions.builder();
+ }
+
+ public final SourceTableColumnInstructions getInstructions(@NotNull final String column) {
+ return columnInstructions().getOrDefault(column, SourceTableColumnInstructions.DEFAULT);
+ }
+
+ public interface Builder {
+ Builder groupingDisabled(boolean disabled);
+
+ Builder putColumnInstructions(@NotNull final String colName,
+ @NotNull final SourceTableColumnInstructions instructions);
+
+ SourceTableInstructions build();
+ }
+}
diff --git a/engine/table/src/main/java/io/deephaven/engine/util/config/MutableInputTable.java b/engine/table/src/main/java/io/deephaven/engine/util/config/MutableInputTable.java
index 202256ca7ea..14e81500f9b 100644
--- a/engine/table/src/main/java/io/deephaven/engine/util/config/MutableInputTable.java
+++ b/engine/table/src/main/java/io/deephaven/engine/util/config/MutableInputTable.java
@@ -4,6 +4,7 @@
package io.deephaven.engine.util.config;
import io.deephaven.engine.exceptions.ArgumentException;
+import io.deephaven.engine.exceptions.IncompatibleTableDefinitionException;
import io.deephaven.engine.table.ColumnDefinition;
import io.deephaven.engine.table.Table;
import io.deephaven.engine.table.TableDefinition;
@@ -52,7 +53,7 @@ default List getValueNames() {
* Helper to check if a table is compatible with this table, so that it could be added as contents.
*
* @param tableToApply the table to check if it can used to add or modify this input table
- * @throws TableDefinition.IncompatibleTableDefinitionException if the definitions are not compatible
+ * @throws IncompatibleTableDefinitionException if the definitions are not compatible
*/
default void validateAddOrModify(final Table tableToApply) {
getTableDefinition().checkMutualCompatibility(tableToApply.getDefinition());
diff --git a/engine/table/src/test/java/io/deephaven/engine/table/impl/TestPartitioningColumns.java b/engine/table/src/test/java/io/deephaven/engine/table/impl/TestPartitioningColumns.java
index 06ac1240ee7..a05af1979a0 100644
--- a/engine/table/src/test/java/io/deephaven/engine/table/impl/TestPartitioningColumns.java
+++ b/engine/table/src/test/java/io/deephaven/engine/table/impl/TestPartitioningColumns.java
@@ -4,25 +4,18 @@
package io.deephaven.engine.table.impl;
import io.deephaven.api.filter.Filter;
-import io.deephaven.chunk.attributes.Values;
import io.deephaven.engine.table.ColumnDefinition;
import io.deephaven.engine.table.Table;
import io.deephaven.engine.table.TableDefinition;
import io.deephaven.engine.testutil.TstUtils;
import io.deephaven.engine.testutil.junit4.EngineCleanup;
import io.deephaven.time.DateTimeUtils;
-import io.deephaven.engine.table.impl.locations.ColumnLocation;
-import io.deephaven.engine.table.impl.locations.TableKey;
-import io.deephaven.engine.table.impl.locations.TableLocation;
-import io.deephaven.engine.table.impl.locations.TableLocationKey;
import io.deephaven.engine.table.impl.locations.impl.*;
import io.deephaven.engine.table.impl.select.MatchFilter;
import io.deephaven.engine.table.impl.select.WhereFilter;
import io.deephaven.engine.table.ColumnSource;
import io.deephaven.engine.table.impl.sources.regioned.*;
import io.deephaven.engine.rowset.RowSetFactory;
-import org.jetbrains.annotations.NotNull;
-import org.jetbrains.annotations.Nullable;
import org.junit.Rule;
import org.junit.Test;
@@ -74,7 +67,7 @@ public void testEverything() {
final TableDefinition resultDefinition = TableDefinition.of(input.getDefinition().getColumnStream()
.map(ColumnDefinition::withPartitioning).collect(Collectors.toList()));
final Table result = new PartitionAwareSourceTable(resultDefinition, "TestPartitioningColumns",
- RegionedTableComponentFactoryImpl.INSTANCE,
+ RegionedTableComponentFactoryImpl.make(),
new PollingTableLocationProvider<>(
StandaloneTableKey.getInstance(),
recordingLocationKeyFinder,
@@ -96,95 +89,4 @@ public void testEverything() {
TstUtils.assertTableEquals(expected.selectDistinct(), result.selectDistinct());
}
-
- private static final class DummyTableLocation extends AbstractTableLocation {
-
- protected DummyTableLocation(@NotNull final TableKey tableKey,
- @NotNull final TableLocationKey tableLocationKey) {
- super(tableKey, tableLocationKey, false);
- }
-
- @Override
- public void refresh() {
-
- }
-
- @NotNull
- @Override
- protected ColumnLocation makeColumnLocation(@NotNull String name) {
- return new ColumnLocation() {
- @NotNull
- @Override
- public TableLocation getTableLocation() {
- return DummyTableLocation.this;
- }
-
- @NotNull
- @Override
- public String getName() {
- return name;
- }
-
- @Override
- public boolean exists() {
- throw new UnsupportedOperationException();
- }
-
- @Nullable
- @Override
- public METADATA_TYPE getMetadata(@NotNull ColumnDefinition> columnDefinition) {
- throw new UnsupportedOperationException();
- }
-
- @Override
- public ColumnRegionChar makeColumnRegionChar(
- @NotNull ColumnDefinition> columnDefinition) {
- throw new UnsupportedOperationException();
- }
-
- @Override
- public ColumnRegionByte makeColumnRegionByte(
- @NotNull ColumnDefinition> columnDefinition) {
- throw new UnsupportedOperationException();
- }
-
- @Override
- public ColumnRegionShort makeColumnRegionShort(
- @NotNull ColumnDefinition> columnDefinition) {
- throw new UnsupportedOperationException();
- }
-
- @Override
- public ColumnRegionInt makeColumnRegionInt(
- @NotNull ColumnDefinition> columnDefinition) {
- throw new UnsupportedOperationException();
- }
-
- @Override
- public ColumnRegionLong makeColumnRegionLong(
- @NotNull ColumnDefinition> columnDefinition) {
- throw new UnsupportedOperationException();
- }
-
- @Override
- public ColumnRegionFloat makeColumnRegionFloat(
- @NotNull ColumnDefinition> columnDefinition) {
- throw new UnsupportedOperationException();
- }
-
- @Override
- public ColumnRegionDouble makeColumnRegionDouble(
- @NotNull ColumnDefinition> columnDefinition) {
- throw new UnsupportedOperationException();
- }
-
- @Override
- public ColumnRegionObject makeColumnRegionObject(
- @NotNull ColumnDefinition columnDefinition) {
- throw new UnsupportedOperationException();
- }
-
- };
- }
- }
}
diff --git a/engine/table/src/test/java/io/deephaven/engine/table/impl/locations/impl/DummyColumnLocation.java b/engine/table/src/test/java/io/deephaven/engine/table/impl/locations/impl/DummyColumnLocation.java
new file mode 100644
index 00000000000..02baff1562b
--- /dev/null
+++ b/engine/table/src/test/java/io/deephaven/engine/table/impl/locations/impl/DummyColumnLocation.java
@@ -0,0 +1,94 @@
+/**
+ * Copyright (c) 2016-2023 Deephaven Data Labs and Patent Pending
+ */
+package io.deephaven.engine.table.impl.locations.impl;
+
+import io.deephaven.chunk.attributes.Values;
+import io.deephaven.engine.table.ColumnDefinition;
+import io.deephaven.engine.table.impl.locations.ColumnLocation;
+import io.deephaven.engine.table.impl.locations.TableLocation;
+import io.deephaven.engine.table.impl.sources.regioned.*;
+import org.jetbrains.annotations.NotNull;
+import org.jetbrains.annotations.Nullable;
+
+public class DummyColumnLocation implements ColumnLocation {
+ private final DummyTableLocation dummyTableLocation;
+ private final @NotNull String name;
+
+ public DummyColumnLocation(DummyTableLocation dummyTableLocation, @NotNull String name) {
+ this.dummyTableLocation = dummyTableLocation;
+ this.name = name;
+ }
+
+ @NotNull
+ @Override
+ public TableLocation getTableLocation() {
+ return dummyTableLocation;
+ }
+
+ @NotNull
+ @Override
+ public String getName() {
+ return name;
+ }
+
+ @Override
+ public boolean exists() {
+ throw new UnsupportedOperationException();
+ }
+
+ @Nullable
+ @Override
+ public METADATA_TYPE getMetadata(@NotNull ColumnDefinition> columnDefinition) {
+ throw new UnsupportedOperationException();
+ }
+
+ @Override
+ public ColumnRegionChar makeColumnRegionChar(
+ @NotNull ColumnDefinition> columnDefinition) {
+ throw new UnsupportedOperationException();
+ }
+
+ @Override
+ public ColumnRegionByte makeColumnRegionByte(
+ @NotNull ColumnDefinition> columnDefinition) {
+ throw new UnsupportedOperationException();
+ }
+
+ @Override
+ public ColumnRegionShort makeColumnRegionShort(
+ @NotNull ColumnDefinition> columnDefinition) {
+ throw new UnsupportedOperationException();
+ }
+
+ @Override
+ public ColumnRegionInt makeColumnRegionInt(
+ @NotNull ColumnDefinition> columnDefinition) {
+ throw new UnsupportedOperationException();
+ }
+
+ @Override
+ public ColumnRegionLong makeColumnRegionLong(
+ @NotNull ColumnDefinition> columnDefinition) {
+ throw new UnsupportedOperationException();
+ }
+
+ @Override
+ public ColumnRegionFloat makeColumnRegionFloat(
+ @NotNull ColumnDefinition> columnDefinition) {
+ throw new UnsupportedOperationException();
+ }
+
+ @Override
+ public ColumnRegionDouble makeColumnRegionDouble(
+ @NotNull ColumnDefinition> columnDefinition) {
+ throw new UnsupportedOperationException();
+ }
+
+ @Override
+ public ColumnRegionObject makeColumnRegionObject(
+ @NotNull ColumnDefinition columnDefinition) {
+ throw new UnsupportedOperationException();
+ }
+
+}
diff --git a/engine/table/src/test/java/io/deephaven/engine/table/impl/locations/impl/DummyTableLocation.java b/engine/table/src/test/java/io/deephaven/engine/table/impl/locations/impl/DummyTableLocation.java
new file mode 100644
index 00000000000..f0d7ee8c82a
--- /dev/null
+++ b/engine/table/src/test/java/io/deephaven/engine/table/impl/locations/impl/DummyTableLocation.java
@@ -0,0 +1,50 @@
+/**
+ * Copyright (c) 2016-2023 Deephaven Data Labs and Patent Pending
+ */
+package io.deephaven.engine.table.impl.locations.impl;
+
+import io.deephaven.api.SortColumn;
+import io.deephaven.engine.table.Table;
+import io.deephaven.engine.table.impl.locations.ColumnLocation;
+import io.deephaven.engine.table.impl.locations.TableKey;
+import io.deephaven.engine.table.impl.locations.TableLocationKey;
+import org.jetbrains.annotations.NotNull;
+import org.jetbrains.annotations.Nullable;
+
+import java.util.List;
+
+public final class DummyTableLocation extends AbstractTableLocation {
+
+ public DummyTableLocation(
+ @NotNull final TableKey tableKey,
+ @NotNull final TableLocationKey tableLocationKey) {
+ super(tableKey, tableLocationKey, false);
+ }
+
+ @Override
+ public void refresh() {
+
+ }
+
+ @Override
+ public @NotNull List getSortedColumns() {
+ // TODO NATE NOCOMMIT?
+ return List.of();
+ }
+
+ @Override
+ public boolean hasDataIndexFor(@NotNull String... columns) {
+ return false;
+ }
+
+ @Override
+ protected @Nullable Table getDataIndexImpl(@NotNull String... columns) {
+ return null;
+ }
+
+ @NotNull
+ @Override
+ protected ColumnLocation makeColumnLocation(@NotNull String name) {
+ return new DummyColumnLocation(this, name);
+ }
+}
diff --git a/engine/table/src/test/java/io/deephaven/engine/table/impl/sources/regioned/TestRegionedColumnSourceManager.java b/engine/table/src/test/java/io/deephaven/engine/table/impl/sources/regioned/TestRegionedColumnSourceManager.java
index 1ad647f0532..be82e2917d1 100644
--- a/engine/table/src/test/java/io/deephaven/engine/table/impl/sources/regioned/TestRegionedColumnSourceManager.java
+++ b/engine/table/src/test/java/io/deephaven/engine/table/impl/sources/regioned/TestRegionedColumnSourceManager.java
@@ -7,6 +7,7 @@
import io.deephaven.engine.rowset.WritableRowSet;
import io.deephaven.engine.table.ColumnDefinition;
import io.deephaven.engine.table.impl.ColumnToCodecMappings;
+import io.deephaven.engine.table.impl.sources.regioned.instructions.SourceTableColumnInstructions;
import io.deephaven.engine.testutil.testcase.RefreshingTableTestCase;
import io.deephaven.engine.table.impl.locations.*;
import io.deephaven.engine.table.impl.locations.impl.SimpleTableLocationKey;
@@ -325,8 +326,10 @@ public Object invoke(Invocation invocation) {
locationIndexToRegionIndex.put(li, regionIndex);
IntStream.range(0, NUM_COLUMNS).forEach(ci -> checking(new Expectations() {
{
- oneOf(columnSources[ci]).addRegion(with(columnDefinitions.get(ci)),
- with(columnLocations[li][ci]));
+ oneOf(columnSources[ci]).addRegion(
+ with(columnDefinitions.get(ci)),
+ with(columnLocations[li][ci]),
+ SourceTableColumnInstructions.DEFAULT);
will(returnValue(regionIndex));
}
}));
diff --git a/engine/table/src/test/java/io/deephaven/engine/table/impl/sources/regioned/TstColumnRegionByte.java b/engine/table/src/test/java/io/deephaven/engine/table/impl/sources/regioned/TstColumnRegionByte.java
index 9e4c4344414..3d75a02549d 100644
--- a/engine/table/src/test/java/io/deephaven/engine/table/impl/sources/regioned/TstColumnRegionByte.java
+++ b/engine/table/src/test/java/io/deephaven/engine/table/impl/sources/regioned/TstColumnRegionByte.java
@@ -4,6 +4,8 @@
package io.deephaven.engine.table.impl.sources.regioned;
import io.deephaven.chunk.attributes.Values;
+import io.deephaven.engine.rowset.WritableRowSet;
+import io.deephaven.engine.table.impl.locations.ColumnLocation;
import io.deephaven.util.QueryConstants;
import io.deephaven.chunk.WritableByteChunk;
import io.deephaven.chunk.WritableChunk;
@@ -62,6 +64,21 @@ public void fillChunkAppend(@NotNull FillContext context, @NotNull WritableChunk
charDestination.setSize(size + length);
}
+
+ @Override
+ public ColumnLocation getLocation() {
+ throw new UnsupportedOperationException();
+ }
+
+ @Override
+ public WritableRowSet match(
+ final boolean invertMatch,
+ final boolean usePrev,
+ final boolean caseInsensitive,
+ @NotNull final RowSequence rowSequence,
+ final Object... sortedKeys) {
+ throw new UnsupportedOperationException("This test region does not support matching");
+ }
}
public static class TestNull extends TstColumnRegionPrimative> {
diff --git a/engine/table/src/test/java/io/deephaven/engine/table/impl/sources/regioned/TstColumnRegionChar.java b/engine/table/src/test/java/io/deephaven/engine/table/impl/sources/regioned/TstColumnRegionChar.java
index a0e2cd9784a..5f184565913 100644
--- a/engine/table/src/test/java/io/deephaven/engine/table/impl/sources/regioned/TstColumnRegionChar.java
+++ b/engine/table/src/test/java/io/deephaven/engine/table/impl/sources/regioned/TstColumnRegionChar.java
@@ -5,6 +5,8 @@
import io.deephaven.chunk.attributes.Values;
import io.deephaven.engine.rowset.RowSequence;
+import io.deephaven.engine.rowset.WritableRowSet;
+import io.deephaven.engine.table.impl.locations.ColumnLocation;
import io.deephaven.util.QueryConstants;
import io.deephaven.chunk.WritableCharChunk;
import io.deephaven.chunk.WritableChunk;
@@ -53,6 +55,21 @@ public void fillChunkAppend(@NotNull FillContext context, @NotNull WritableChunk
charDestination.setSize(size + length);
}
+
+ @Override
+ public ColumnLocation getLocation() {
+ throw new UnsupportedOperationException();
+ }
+
+ @Override
+ public WritableRowSet match(
+ final boolean invertMatch,
+ final boolean usePrev,
+ final boolean caseInsensitive,
+ @NotNull final RowSequence rowSequence,
+ final Object... sortedKeys) {
+ throw new UnsupportedOperationException("This test region does not support matching");
+ }
}
public static class TestNull extends TstColumnRegionPrimative> {
diff --git a/engine/table/src/test/java/io/deephaven/engine/table/impl/sources/regioned/TstColumnRegionDouble.java b/engine/table/src/test/java/io/deephaven/engine/table/impl/sources/regioned/TstColumnRegionDouble.java
index 5ab8dc37aa6..d65bb654fa7 100644
--- a/engine/table/src/test/java/io/deephaven/engine/table/impl/sources/regioned/TstColumnRegionDouble.java
+++ b/engine/table/src/test/java/io/deephaven/engine/table/impl/sources/regioned/TstColumnRegionDouble.java
@@ -10,6 +10,8 @@
import io.deephaven.chunk.attributes.Values;
import io.deephaven.engine.rowset.RowSequence;
+import io.deephaven.engine.rowset.WritableRowSet;
+import io.deephaven.engine.table.impl.locations.ColumnLocation;
import io.deephaven.util.QueryConstants;
import io.deephaven.chunk.WritableDoubleChunk;
import io.deephaven.chunk.WritableChunk;
@@ -58,6 +60,21 @@ public void fillChunkAppend(@NotNull FillContext context, @NotNull WritableChunk
doubleDestination.setSize(size + length);
}
+
+ @Override
+ public ColumnLocation getLocation() {
+ throw new UnsupportedOperationException();
+ }
+
+ @Override
+ public WritableRowSet match(
+ final boolean invertMatch,
+ final boolean usePrev,
+ final boolean caseInsensitive,
+ @NotNull final RowSequence rowSequence,
+ final Object... sortedKeys) {
+ throw new UnsupportedOperationException("This test region does not support matching");
+ }
}
public static class TestNull extends TstColumnRegionPrimative> {
diff --git a/engine/table/src/test/java/io/deephaven/engine/table/impl/sources/regioned/TstColumnRegionFloat.java b/engine/table/src/test/java/io/deephaven/engine/table/impl/sources/regioned/TstColumnRegionFloat.java
index 799630bc19a..338c5d3f1b2 100644
--- a/engine/table/src/test/java/io/deephaven/engine/table/impl/sources/regioned/TstColumnRegionFloat.java
+++ b/engine/table/src/test/java/io/deephaven/engine/table/impl/sources/regioned/TstColumnRegionFloat.java
@@ -10,6 +10,8 @@
import io.deephaven.chunk.attributes.Values;
import io.deephaven.engine.rowset.RowSequence;
+import io.deephaven.engine.rowset.WritableRowSet;
+import io.deephaven.engine.table.impl.locations.ColumnLocation;
import io.deephaven.util.QueryConstants;
import io.deephaven.chunk.WritableFloatChunk;
import io.deephaven.chunk.WritableChunk;
@@ -58,6 +60,21 @@ public void fillChunkAppend(@NotNull FillContext context, @NotNull WritableChunk
floatDestination.setSize(size + length);
}
+
+ @Override
+ public ColumnLocation getLocation() {
+ throw new UnsupportedOperationException();
+ }
+
+ @Override
+ public WritableRowSet match(
+ final boolean invertMatch,
+ final boolean usePrev,
+ final boolean caseInsensitive,
+ @NotNull final RowSequence rowSequence,
+ final Object... sortedKeys) {
+ throw new UnsupportedOperationException("This test region does not support matching");
+ }
}
public static class TestNull extends TstColumnRegionPrimative