From f9709af688420a624167a17f5b9c3de063a144df Mon Sep 17 00:00:00 2001 From: ntisseyre Date: Thu, 10 Oct 2024 14:10:18 -0500 Subject: [PATCH] Draft (Inlined properties into index structure) --- .../janusgraph/graphdb/JanusGraphTest.java | 36 ++++-- .../core/schema/JanusGraphIndex.java | 7 ++ .../core/schema/JanusGraphManagement.java | 9 ++ .../graphdb/database/IndexSerializer.java | 119 ++++++++++++------ .../graphdb/database/StandardJanusGraph.java | 21 ++-- .../database/index/IndexMutationType.java | 1 + .../graphdb/database/index/IndexUpdate.java | 9 +- .../database/index/IndexUpdateContainer.java | 59 +++++++++ .../management/JanusGraphIndexWrapper.java | 9 ++ .../database/management/ManagementSystem.java | 43 ++++++- .../database/util/IndexRecordUtil.java | 106 ++++++++++++++-- .../database/util/StaleIndexRecordUtil.java | 10 +- .../internal/InternalRelationType.java | 3 + .../graphdb/olap/job/IndexRepairJob.java | 2 +- .../query/graph/GraphCentricQueryBuilder.java | 4 +- .../query/vertex/VertexWithInlineProps.java | 89 +++++++++++++ .../transaction/StandardJanusGraphTx.java | 15 ++- .../graphdb/types/CompositeIndexType.java | 4 + .../graphdb/types/TypeDefinitionCategory.java | 3 +- .../indextype/CompositeIndexTypeWrapper.java | 22 ++++ .../types/indextype/IndexReferenceType.java | 37 ++++++ .../graphdb/types/system/BaseKey.java | 12 ++ .../types/system/EmptyRelationType.java | 6 + .../types/vertices/RelationTypeVertex.java | 61 +++++++-- .../graphdb/util/SubqueryIterator.java | 7 +- .../graphdb/vertices/CacheVertex.java | 2 +- .../graphdb/database/IndexSerializerTest.java | 6 +- 27 files changed, 599 insertions(+), 103 deletions(-) create mode 100644 janusgraph-core/src/main/java/org/janusgraph/graphdb/database/index/IndexUpdateContainer.java create mode 100644 janusgraph-core/src/main/java/org/janusgraph/graphdb/query/vertex/VertexWithInlineProps.java create mode 100644 janusgraph-core/src/main/java/org/janusgraph/graphdb/types/indextype/IndexReferenceType.java diff --git a/janusgraph-backend-testutils/src/main/java/org/janusgraph/graphdb/JanusGraphTest.java b/janusgraph-backend-testutils/src/main/java/org/janusgraph/graphdb/JanusGraphTest.java index 717451ab1f..b604fe4ad2 100644 --- a/janusgraph-backend-testutils/src/main/java/org/janusgraph/graphdb/JanusGraphTest.java +++ b/janusgraph-backend-testutils/src/main/java/org/janusgraph/graphdb/JanusGraphTest.java @@ -9083,6 +9083,7 @@ public void testDirectCompositeIndexEntryModification() throws BackendException IndexRecordEntry[] record = new IndexRecordEntry[]{new IndexRecordEntry(indexedProperty)}; JanusGraphElement element = (JanusGraphElement) vertex1; Serializer serializer = graph.getDataSerializer(); + EdgeSerializer edgeSerializer = graph.getEdgeSerializer(); boolean hashKeys = graph.getIndexSerializer().isHashKeys(); HashingUtil.HashLength hashLength = graph.getIndexSerializer().getHashLength(); @@ -9092,6 +9093,8 @@ public void testDirectCompositeIndexEntryModification() throws BackendException record, element, serializer, + (StandardJanusGraphTx)tx, + edgeSerializer, hashKeys, hashLength ); @@ -9116,6 +9119,8 @@ public void testDirectCompositeIndexEntryModification() throws BackendException record, element, serializer, + (StandardJanusGraphTx)tx, + edgeSerializer, hashKeys, hashLength ); @@ -9150,6 +9155,8 @@ record = new IndexRecordEntry[]{new IndexRecordEntry(propertyId, "vertex2", prop record, element, serializer, + (StandardJanusGraphTx)tx, + edgeSerializer, hashKeys, hashLength ); @@ -9176,6 +9183,8 @@ record = new IndexRecordEntry[]{new IndexRecordEntry(propertyKey.longId(), "vert record, element, serializer, + (StandardJanusGraphTx)tx, + edgeSerializer, hashKeys, hashLength ); @@ -9218,6 +9227,7 @@ public void testStaleIndexForceRemoveVertexFromGraphIndex() throws BackendExcept JanusGraphSchemaVertex indexChangeVertex = managementSystem.getSchemaVertex(janusGraphIndex); CompositeIndexType index = (CompositeIndexType) indexChangeVertex.asIndexType(); Serializer serializer = graph.getDataSerializer(); + EdgeSerializer edgeSerializer = graph.getEdgeSerializer(); boolean hashKeys = graph.getIndexSerializer().isHashKeys(); HashingUtil.HashLength hashLength = graph.getIndexSerializer().getHashLength(); @@ -9238,6 +9248,8 @@ public void testStaleIndexForceRemoveVertexFromGraphIndex() throws BackendExcept record, element, serializer, + (StandardJanusGraphTx)tx, + edgeSerializer, hashKeys, hashLength ); @@ -9301,6 +9313,7 @@ public void testEdgeEntryIndexForceRemoveFromGraphIndex() throws BackendExceptio JanusGraphSchemaVertex indexChangeVertex = managementSystem.getSchemaVertex(janusGraphIndex); CompositeIndexType index = (CompositeIndexType) indexChangeVertex.asIndexType(); Serializer serializer = graph.getDataSerializer(); + EdgeSerializer edgeSerializer = graph.getEdgeSerializer(); boolean hashKeys = graph.getIndexSerializer().isHashKeys(); HashingUtil.HashLength hashLength = graph.getIndexSerializer().getHashLength(); PropertyKey propertyKey = managementSystem.getPropertyKey(namePropKeyStr); @@ -9334,6 +9347,8 @@ public void testEdgeEntryIndexForceRemoveFromGraphIndex() throws BackendExceptio record, element, serializer, + (StandardJanusGraphTx)tx, + edgeSerializer, hashKeys, hashLength ); @@ -9394,6 +9409,7 @@ public void testStaleIndexForceRemoveVertexFromGraphIndexByHelperMethod(boolean JanusGraphSchemaVertex indexChangeVertex = managementSystem.getSchemaVertex(janusGraphIndex); CompositeIndexType index = (CompositeIndexType) indexChangeVertex.asIndexType(); Serializer serializer = graph.getDataSerializer(); + EdgeSerializer edgeSerializer = graph.getEdgeSerializer(); boolean hashKeys = graph.getIndexSerializer().isHashKeys(); HashingUtil.HashLength hashLength = graph.getIndexSerializer().getHashLength(); @@ -9422,6 +9438,8 @@ public void testStaleIndexForceRemoveVertexFromGraphIndexByHelperMethod(boolean record, element, serializer, + (StandardJanusGraphTx)tx, + edgeSerializer, hashKeys, hashLength ); @@ -10075,28 +10093,28 @@ private void invalidateUpdatedVertexProperty(StandardJanusGraph graph, Object ve invalidateUpdatedVertexProperty(graph, vertexIdUpdated, propertyNameUpdated, previousPropertyValue, newPropertyValue, true); } - private void invalidateUpdatedVertexProperty(StandardJanusGraph graph, Object vertexIdUpdated, String propertyNameUpdated, Object previousPropertyValue, Object newPropertyValue, boolean withIndexConstraintsFilter){ - JanusGraphTransaction tx = graph.newTransaction(); + private void invalidateUpdatedVertexProperty(StandardJanusGraph graph, Object vertexIdUpdated, String propertyNameUpdated, Object previousPropertyValue, Object newPropertyValue, boolean withIndexConstraintsFilter) { + StandardJanusGraphTx tx = (StandardJanusGraphTx) graph.newTransaction(); JanusGraphManagement graphMgmt = graph.openManagement(); PropertyKey propertyKey = graphMgmt.getPropertyKey(propertyNameUpdated); - CacheVertex cacheVertex = new CacheVertex((StandardJanusGraphTx) tx, vertexIdUpdated, ElementLifeCycle.Loaded); + CacheVertex cacheVertex = new CacheVertex(tx, vertexIdUpdated, ElementLifeCycle.Loaded); StandardVertexProperty propertyPreviousVal = new StandardVertexProperty(propertyKey.longId(), propertyKey, cacheVertex, previousPropertyValue, ElementLifeCycle.Removed); StandardVertexProperty propertyNewVal = new StandardVertexProperty(propertyKey.longId(), propertyKey, cacheVertex, newPropertyValue, ElementLifeCycle.New); IndexSerializer indexSerializer = graph.getIndexSerializer(); - Collection indexUpdates; - if(withIndexConstraintsFilter){ - indexUpdates = indexSerializer.getIndexUpdates(cacheVertex, Arrays.asList(propertyPreviousVal, propertyNewVal)); + Stream indexUpdates; + if (withIndexConstraintsFilter) { + indexUpdates = indexSerializer.getIndexUpdates(cacheVertex, Arrays.asList(propertyPreviousVal, propertyNewVal), tx); } else { - indexUpdates = indexSerializer.getIndexUpdatesNoConstraints(cacheVertex, Arrays.asList(propertyPreviousVal, propertyNewVal)); + indexUpdates = indexSerializer.getIndexUpdatesNoConstraints(cacheVertex, Arrays.asList(propertyPreviousVal, propertyNewVal), tx); } CacheInvalidationService invalidationService = graph.getDBCacheInvalidationService(); - for(IndexUpdate indexUpdate : indexUpdates){ + indexUpdates.forEach(indexUpdate -> { StaticBuffer keyToInvalidate = (StaticBuffer) indexUpdate.getKey(); invalidationService.markKeyAsExpiredInIndexStore(keyToInvalidate); - } + }); invalidationService.forceClearExpiredKeysInIndexStoreCache(); invalidationService.forceInvalidateVertexInEdgeStoreCache(vertexIdUpdated); diff --git a/janusgraph-core/src/main/java/org/janusgraph/core/schema/JanusGraphIndex.java b/janusgraph-core/src/main/java/org/janusgraph/core/schema/JanusGraphIndex.java index e26605b17d..67fdac2339 100644 --- a/janusgraph-core/src/main/java/org/janusgraph/core/schema/JanusGraphIndex.java +++ b/janusgraph-core/src/main/java/org/janusgraph/core/schema/JanusGraphIndex.java @@ -83,6 +83,13 @@ default Object id() { */ PropertyKey[] getFieldKeys(); + /** + * Returns the inlined keys of this index. + * + * @return + */ + String[] getInlineFieldKeys(); + /** * Returns the parameters associated with an indexed key of this index. Parameters modify the indexing * behavior of the underlying indexing backend. diff --git a/janusgraph-core/src/main/java/org/janusgraph/core/schema/JanusGraphManagement.java b/janusgraph-core/src/main/java/org/janusgraph/core/schema/JanusGraphManagement.java index 43d8047612..a2fc563e89 100644 --- a/janusgraph-core/src/main/java/org/janusgraph/core/schema/JanusGraphManagement.java +++ b/janusgraph-core/src/main/java/org/janusgraph/core/schema/JanusGraphManagement.java @@ -182,6 +182,8 @@ public interface JanusGraphManagement extends JanusGraphConfiguration, SchemaMan void addIndexKey(final JanusGraphIndex index, final PropertyKey key, Parameter... parameters); + void addInlinePropertyKey(final JanusGraphIndex index, final PropertyKey key); + /** * Builder for {@link JanusGraphIndex}. Allows for the configuration of a graph index prior to its construction. */ @@ -195,6 +197,13 @@ interface IndexBuilder { */ IndexBuilder addKey(PropertyKey key); + /** + * Adds the given key to inline properties of the composite key of this index + * @param key + * @return this IndexBuilder + */ + IndexBuilder addInlinePropertyKey(PropertyKey key); + /** * Adds the given key and associated parameters to the composite key of this index * diff --git a/janusgraph-core/src/main/java/org/janusgraph/graphdb/database/IndexSerializer.java b/janusgraph-core/src/main/java/org/janusgraph/graphdb/database/IndexSerializer.java index 3ca1b49c47..1c84d5426b 100644 --- a/janusgraph-core/src/main/java/org/janusgraph/graphdb/database/IndexSerializer.java +++ b/janusgraph-core/src/main/java/org/janusgraph/graphdb/database/IndexSerializer.java @@ -43,13 +43,17 @@ import org.janusgraph.diskstorage.indexing.RawQuery; import org.janusgraph.diskstorage.indexing.StandardKeyInformation; import org.janusgraph.diskstorage.keycolumnvalue.KeySliceQuery; +import org.janusgraph.diskstorage.keycolumnvalue.SliceQuery; import org.janusgraph.diskstorage.util.BufferUtil; +import org.janusgraph.diskstorage.util.EntryArrayList; import org.janusgraph.diskstorage.util.HashingUtil; import org.janusgraph.graphdb.database.idhandling.IDHandler; +import org.janusgraph.graphdb.database.idhandling.VariableLong; import org.janusgraph.graphdb.database.index.IndexInfoRetriever; import org.janusgraph.graphdb.database.index.IndexMutationType; import org.janusgraph.graphdb.database.index.IndexRecords; import org.janusgraph.graphdb.database.index.IndexUpdate; +import org.janusgraph.graphdb.database.index.IndexUpdateContainer; import org.janusgraph.graphdb.database.serialize.Serializer; import org.janusgraph.graphdb.database.util.IndexAppliesToFunction; import org.janusgraph.graphdb.database.util.IndexRecordUtil; @@ -66,6 +70,7 @@ import org.janusgraph.graphdb.query.graph.JointIndexQuery; import org.janusgraph.graphdb.query.graph.MultiKeySliceQuery; import org.janusgraph.graphdb.query.index.IndexSelectionUtil; +import org.janusgraph.graphdb.query.vertex.VertexWithInlineProps; import org.janusgraph.graphdb.relations.RelationIdentifier; import org.janusgraph.graphdb.tinkerpop.optimize.step.Aggregation; import org.janusgraph.graphdb.transaction.StandardJanusGraphTx; @@ -74,6 +79,8 @@ import org.janusgraph.graphdb.types.MixedIndexType; import org.janusgraph.graphdb.types.ParameterIndexField; import org.janusgraph.graphdb.types.ParameterType; +import org.janusgraph.graphdb.types.TypeInspector; +import org.janusgraph.graphdb.types.indextype.IndexReferenceType; import org.slf4j.Logger; import org.slf4j.LoggerFactory; @@ -114,6 +121,7 @@ public class IndexSerializer { private static final Logger log = LoggerFactory.getLogger(IndexSerializer.class); + private final EdgeSerializer edgeSerializer; private final Serializer serializer; private final Configuration configuration; private final Map mixedIndexes; @@ -121,8 +129,13 @@ public class IndexSerializer { private final boolean hashKeys; private final HashingUtil.HashLength hashLength = HashingUtil.HashLength.SHORT; - public IndexSerializer(Configuration config, Serializer serializer, Map indexes, final boolean hashKeys) { + public IndexSerializer(Configuration config, + EdgeSerializer edgeSerializer, + Serializer serializer, + Map indexes, + final boolean hashKeys) { this.serializer = serializer; + this.edgeSerializer = edgeSerializer; this.configuration = config; this.mixedIndexes = indexes; this.hashKeys=hashKeys; @@ -186,27 +199,27 @@ public IndexInfoRetriever getIndexInfoRetriever(StandardJanusGraphTx tx) { Index Updates ################################################### */ - public Collection getIndexUpdates(InternalRelation relation) { - return getIndexUpdates(relation, FULL_INDEX_APPLIES_TO_FILTER); + public Collection getIndexUpdates(InternalRelation relation, TypeInspector typeInspector) { + return getIndexUpdates(relation, FULL_INDEX_APPLIES_TO_FILTER, typeInspector); } - public Collection getIndexUpdates(InternalVertex vertex, Collection updatedProperties) { - return getIndexUpdates(vertex, updatedProperties, FULL_INDEX_APPLIES_TO_FILTER); + public Stream getIndexUpdates(InternalVertex vertex, Collection updatedProperties, TypeInspector typeInspector) { + return getIndexUpdates(vertex, updatedProperties, FULL_INDEX_APPLIES_TO_FILTER, typeInspector); } - public Collection getIndexUpdatesNoConstraints(InternalRelation relation) { - return getIndexUpdates(relation, INDEX_APPLIES_TO_NO_CONSTRAINTS_FILTER); + public Collection getIndexUpdatesNoConstraints(InternalRelation relation, TypeInspector typeInspector) { + return getIndexUpdates(relation, INDEX_APPLIES_TO_NO_CONSTRAINTS_FILTER, typeInspector); } - public Collection getIndexUpdatesNoConstraints(InternalVertex vertex, Collection updatedProperties) { - return getIndexUpdates(vertex, updatedProperties, INDEX_APPLIES_TO_NO_CONSTRAINTS_FILTER); + public Stream getIndexUpdatesNoConstraints(InternalVertex vertex, Collection updatedProperties, TypeInspector typeInspector) { + return getIndexUpdates(vertex, updatedProperties, INDEX_APPLIES_TO_NO_CONSTRAINTS_FILTER, typeInspector); } - public Collection getIndexUpdates(InternalRelation relation, IndexAppliesToFunction indexFilter) { + public Collection getIndexUpdates(InternalRelation relation, IndexAppliesToFunction indexFilter, TypeInspector typeInspector) { assert relation.isNew() || relation.isRemoved(); final Set updates = new HashSet<>(); - final IndexMutationType updateType = getUpdateType(relation); - final int ttl = updateType==IndexMutationType.ADD?StandardJanusGraph.getTTL(relation):0; + final IndexMutationType updateType = getUpdateType(relation, false); + final int ttl = updateType == IndexMutationType.DELETE ? 0: StandardJanusGraph.getTTL(relation); for (final PropertyKey type : relation.getPropertyKeysDirect()) { if (type == null) continue; for (final IndexType index : ((InternalRelationType) type).getKeyIndexes()) { @@ -216,7 +229,7 @@ public Collection getIndexUpdates(InternalRelation relation, IndexA final CompositeIndexType iIndex= (CompositeIndexType) index; final IndexRecordEntry[] record = indexMatch(relation, iIndex); if (record==null) continue; - update = getCompositeIndexUpdate(iIndex, updateType, record, relation, serializer, hashKeys, hashLength); + update = getCompositeIndexUpdate(iIndex, updateType, record, relation, serializer, typeInspector, edgeSerializer, hashKeys, hashLength); } else { assert relation.valueOrNull(type)!=null; if (((MixedIndexType)index).getField(type).getStatus()== SchemaStatus.DISABLED) continue; @@ -229,40 +242,58 @@ public Collection getIndexUpdates(InternalRelation relation, IndexA return updates; } - public Collection getIndexUpdates(InternalVertex vertex, Collection updatedProperties, IndexAppliesToFunction indexFilter) { - if (updatedProperties.isEmpty()) return Collections.emptyList(); - final Set updates = new HashSet<>(); + public Stream getIndexUpdates(InternalVertex vertex, + Collection updatedProperties, + IndexAppliesToFunction indexFilter, + TypeInspector typeInspector) { + + if (updatedProperties.isEmpty()) return Stream.empty(); + final Map updates = new HashMap<>(); for (final InternalRelation rel : updatedProperties) { assert rel.isProperty(); final JanusGraphVertexProperty p = (JanusGraphVertexProperty)rel; assert rel.isNew() || rel.isRemoved(); assert rel.getVertex(0).equals(vertex); - final IndexMutationType updateType = getUpdateType(rel); - for (final IndexType index : ((InternalRelationType)p.propertyKey()).getKeyIndexes()) { - if (!indexFilter.indexAppliesTo(index,vertex)) continue; - if (index.isCompositeIndex()) { //Gather composite indexes - final CompositeIndexType cIndex = (CompositeIndexType)index; - final IndexRecords updateRecords = indexMatches(vertex,cIndex,updateType==IndexMutationType.DELETE,p.propertyKey(),new IndexRecordEntry(p)); + + for (final IndexReferenceType indexRef : ((InternalRelationType) p.propertyKey()).getKeyIndexesReferences()) { + final IndexMutationType updateType = getUpdateType(rel, indexRef.isInlined()); + + if (vertex.isRemoved() && indexRef.isInlined()) continue; + if (!indexFilter.indexAppliesTo(indexRef.getIndexType(), vertex)) continue; + + if (indexRef.getIndexType().isCompositeIndex()) { //Gather composite indexes + final CompositeIndexType cIndex = (CompositeIndexType) indexRef.getIndexType(); + final IndexRecords updateRecords = indexMatches(vertex,cIndex, rel.isRemoved(), p.propertyKey(), new IndexRecordEntry(p)); for (final IndexRecordEntry[] record : updateRecords) { - final IndexUpdate update = getCompositeIndexUpdate(cIndex, updateType, record, vertex, serializer, hashKeys, hashLength); - final int ttl = getIndexTTL(vertex,getKeysOfRecords(record)); - if (ttl>0 && updateType== IndexMutationType.ADD) update.setTTL(ttl); - updates.add(update); + final IndexUpdate update = getCompositeIndexUpdate(cIndex, updateType, record, vertex, serializer, typeInspector, edgeSerializer, hashKeys, hashLength); + final int ttl = getIndexTTL(vertex, getKeysOfRecords(record)); + if (ttl > 0 && updateType != IndexMutationType.DELETE) update.setTTL(ttl); + if (updates.containsKey(update.getKey())) { + updates.get(update.getKey()).add(update); + } else { + updates.put(update.getKey(), new IndexUpdateContainer(update)); + } } } else { //Update mixed indexes - ParameterIndexField field = ((MixedIndexType)index).getField(p.propertyKey()); + ParameterIndexField field = ((MixedIndexType) indexRef.getIndexType()).getField(p.propertyKey()); if (field == null) { - throw new SchemaViolationException(p.propertyKey() + " is not available in mixed index " + index); + throw new SchemaViolationException(p.propertyKey() + " is not available in mixed index " + indexRef.getIndexType()); } if (field.getStatus() == SchemaStatus.DISABLED) continue; - final IndexUpdate update = getMixedIndexUpdate(vertex, p.propertyKey(), p.value(), (MixedIndexType) index, updateType); - final int ttl = getIndexTTL(vertex,p.propertyKey()); - if (ttl>0 && updateType== IndexMutationType.ADD) update.setTTL(ttl); - updates.add(update); + final IndexUpdate update = getMixedIndexUpdate(vertex, p.propertyKey(), p.value(), (MixedIndexType) indexRef.getIndexType(), updateType); + final int ttl = getIndexTTL(vertex, p.propertyKey()); + + if (ttl>0 && updateType != IndexMutationType.DELETE) update.setTTL(ttl); + if (updates.containsKey(update.getKey())) { + updates.get(update.getKey()).add(update); + } else { + updates.put(update.getKey(), new IndexUpdateContainer(update)); + } } } } - return updates; + + return updates.values().stream().flatMap(IndexUpdateContainer::getUpdates); } public boolean reindexElement(JanusGraphElement element, MixedIndexType index, Map>> documentsPerStore) { @@ -292,7 +323,7 @@ public void removeElement(Object elementId, MixedIndexType index, Map()); } - public Set> reindexElement(JanusGraphElement element, CompositeIndexType index) { + public Set> reindexElement(JanusGraphElement element, CompositeIndexType index, TypeInspector typeInspector) { final Set> indexEntries = new HashSet<>(); if (!indexAppliesTo(index,element)) { return indexEntries; @@ -306,7 +337,8 @@ public Set> reindexElement(JanusGraphElement ele records = (record == null) ? Collections.emptyList() : Collections.singletonList(record); } for (final IndexRecordEntry[] record : records) { - indexEntries.add(getCompositeIndexUpdate(index, IndexMutationType.ADD, record, element, serializer, hashKeys, hashLength)); + indexEntries.add(getCompositeIndexUpdate(index, IndexMutationType.ADD, record, element, serializer, + typeInspector, edgeSerializer, hashKeys, hashLength)); } return indexEntries; } @@ -315,23 +347,28 @@ public Set> reindexElement(JanusGraphElement ele Querying ################################################### */ - public Stream query(final JointIndexQuery.Subquery query, final BackendTransaction tx) { + public Stream query(final JointIndexQuery.Subquery query, final BackendTransaction tx, StandardJanusGraphTx standardJanusGraphTx) { final IndexType index = query.getIndex(); if (index.isCompositeIndex()) { + Map inlineQueries = IndexRecordUtil.getInlinePropertiesQueries((CompositeIndexType) index, standardJanusGraphTx); final MultiKeySliceQuery sq = query.getCompositeQuery(); final List rs = sq.execute(tx); final List results = new ArrayList<>(rs.get(0).size()); for (final EntryList r : rs) { for (final java.util.Iterator iterator = r.reuseIterator(); iterator.hasNext(); ) { final Entry entry = iterator.next(); - final ReadBuffer entryValue = entry.asReadBuffer(); - entryValue.movePositionTo(entry.getValuePosition()); - switch(index.getElement()) { + final ReadBuffer readBuffer = entry.asReadBuffer(); + readBuffer.movePositionTo(entry.getValuePosition()); + switch (index.getElement()) { case VERTEX: - results.add(IDHandler.readVertexId(entryValue, true)); + Object vertexId = IDHandler.readVertexId(readBuffer, true); + results.add(new VertexWithInlineProps(vertexId, + EntryArrayList.of(IndexRecordUtil.readInlineProperties(readBuffer)), + inlineQueries, + standardJanusGraphTx)); break; default: - results.add(bytebuffer2RelationId(entryValue)); + results.add(bytebuffer2RelationId(readBuffer)); } } } diff --git a/janusgraph-core/src/main/java/org/janusgraph/graphdb/database/StandardJanusGraph.java b/janusgraph-core/src/main/java/org/janusgraph/graphdb/database/StandardJanusGraph.java index d4f8bc4292..66514489e0 100644 --- a/janusgraph-core/src/main/java/org/janusgraph/graphdb/database/StandardJanusGraph.java +++ b/janusgraph-core/src/main/java/org/janusgraph/graphdb/database/StandardJanusGraph.java @@ -218,10 +218,10 @@ public StandardJanusGraph(GraphDatabaseConfiguration configuration) { backend.getEdgeStoreCache(), backend.getIndexStoreCache(), idManager); this.serializer = config.getSerializer(); + this.edgeSerializer = new EdgeSerializer(this.serializer); StoreFeatures storeFeatures = backend.getStoreFeatures(); - this.indexSerializer = new IndexSerializer(configuration.getConfiguration(), this.serializer, + this.indexSerializer = new IndexSerializer(configuration.getConfiguration(), this.edgeSerializer, this.serializer, this.backend.getIndexInformation(), storeFeatures.isDistributed() && storeFeatures.isKeyOrdered()); - this.edgeSerializer = new EdgeSerializer(this.serializer); this.vertexExistenceQuery = edgeSerializer.getQuery(BaseKey.VertexExists, Direction.OUT, new EdgeSerializer.TypedInterval[0]).setLimit(1); this.queryCache = new RelationQueryCache(this.edgeSerializer); this.schemaCache = configuration.getTypeCache(typeCacheRetrieval); @@ -719,7 +719,7 @@ public ModificationSummary prepareCommit(final Collection adde if (isBigDataSetLoggingEnabled) { logForPrepareCommit.debug("3. Collect all index update for vertices"); } - prepareCommitVertexIndexUpdates(mutatedProperties, indexUpdates); + prepareCommitVertexIndexUpdates(mutatedProperties, tx, indexUpdates); if (isBigDataSetLoggingEnabled) { logForPrepareCommit.debug("4. Acquire index locks (deletions first)"); @@ -770,7 +770,7 @@ private void prepareCommitDeletes(final Collection deletedRela mutator.acquireEdgeLock(idManager.getKey(vertex.id()), entry); } } - indexUpdates.addAll(indexSerializer.getIndexUpdates(del)); + indexUpdates.addAll(indexSerializer.getIndexUpdates(del, tx)); } } @@ -801,7 +801,7 @@ private void prepareCommitAdditions(final Collection addedRela mutator.acquireEdgeLock(idManager.getKey(vertex.id()), entry.getColumn()); } } - indexUpdates.addAll(indexSerializer.getIndexUpdates(add)); + indexUpdates.addAll(indexSerializer.getIndexUpdates(add, tx)); } } @@ -809,14 +809,11 @@ private void prepareCommitAdditions(final Collection addedRela * Collect all index update for vertices */ private void prepareCommitVertexIndexUpdates(final ListMultimap mutatedProperties, + final StandardJanusGraphTx tx, final List indexUpdates) { - mutatedProperties.keySet().parallelStream() - .map(v -> indexSerializer.getIndexUpdates(v, mutatedProperties.get(v))) - // Note: due to usage of parallel stream, the collector is used to synchronize insertions - // into `indexUpdates` for thread safety reasons. - // Using `forEach` directly isn't thread safe. - .collect(Collectors.toList()) - .forEach(indexUpdates::addAll); + indexUpdates.addAll(mutatedProperties.keySet().parallelStream() + .flatMap(v -> indexSerializer.getIndexUpdates(v, mutatedProperties.get(v), tx)) + .collect(Collectors.toList())); } /** diff --git a/janusgraph-core/src/main/java/org/janusgraph/graphdb/database/index/IndexMutationType.java b/janusgraph-core/src/main/java/org/janusgraph/graphdb/database/index/IndexMutationType.java index ad21bbb5ad..13ba183638 100644 --- a/janusgraph-core/src/main/java/org/janusgraph/graphdb/database/index/IndexMutationType.java +++ b/janusgraph-core/src/main/java/org/janusgraph/graphdb/database/index/IndexMutationType.java @@ -16,5 +16,6 @@ public enum IndexMutationType { ADD, + UPDATE, DELETE } diff --git a/janusgraph-core/src/main/java/org/janusgraph/graphdb/database/index/IndexUpdate.java b/janusgraph-core/src/main/java/org/janusgraph/graphdb/database/index/IndexUpdate.java index d595e17a43..84d3d5f0de 100644 --- a/janusgraph-core/src/main/java/org/janusgraph/graphdb/database/index/IndexUpdate.java +++ b/janusgraph-core/src/main/java/org/janusgraph/graphdb/database/index/IndexUpdate.java @@ -65,9 +65,12 @@ public E getEntry() { } public boolean isAddition() { - return mutationType== IndexMutationType.ADD; + return !isDeletion(); } + public boolean isUpdate() { + return mutationType == IndexMutationType.UPDATE; + } public boolean isDeletion() { return mutationType== IndexMutationType.DELETE; } @@ -81,8 +84,8 @@ public boolean isMixedIndex() { } public void setTTL(int ttl) { - Preconditions.checkArgument(ttl>0 && mutationType == IndexMutationType.ADD); - ((MetaAnnotatable)entry).setMetaData(EntryMetaData.TTL,ttl); + Preconditions.checkArgument(ttl > 0 && mutationType != IndexMutationType.DELETE); + ((MetaAnnotatable) entry).setMetaData(EntryMetaData.TTL, ttl); } @Override diff --git a/janusgraph-core/src/main/java/org/janusgraph/graphdb/database/index/IndexUpdateContainer.java b/janusgraph-core/src/main/java/org/janusgraph/graphdb/database/index/IndexUpdateContainer.java new file mode 100644 index 0000000000..427d4f1545 --- /dev/null +++ b/janusgraph-core/src/main/java/org/janusgraph/graphdb/database/index/IndexUpdateContainer.java @@ -0,0 +1,59 @@ +// Copyright 2022 Unified Catalog Team +// +// Licensed under the Apache License, Version 2.0 (the "License"); +// you may not use this file except in compliance with the License. +// You may obtain a copy of the License at +// +// http://www.apache.org/licenses/LICENSE-2.0 +// +// Unless required by applicable law or agreed to in writing, software +// distributed under the License is distributed on an "AS IS" BASIS, +// WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. +// See the License for the specific language governing permissions and +// limitations under the License. + +package org.janusgraph.graphdb.database.index; + +import com.google.common.collect.Sets; + +import java.util.Set; +import java.util.stream.Stream; + +public class IndexUpdateContainer { + + private Set addDelete = null; + + private IndexUpdate updateOnly = null; + + public IndexUpdateContainer(IndexUpdate indexUpdate) { + if (indexUpdate.isUpdate()) { + updateOnly = indexUpdate; + } else { + initSet(indexUpdate); + } + } + + public void add(IndexUpdate indexUpdate) { + if (!indexUpdate.isUpdate()) { + initSet(indexUpdate); + } + } + + public Stream getUpdates() { + if (updateOnly != null) { + return Stream.of(updateOnly); + } else { + return this.addDelete.stream(); + } + } + + private void initSet(IndexUpdate indexUpdate) { + if (this.addDelete == null) { + this.addDelete = Sets.newHashSet(); + } + + this.addDelete.add(indexUpdate); + updateOnly = null; + } + +} diff --git a/janusgraph-core/src/main/java/org/janusgraph/graphdb/database/management/JanusGraphIndexWrapper.java b/janusgraph-core/src/main/java/org/janusgraph/graphdb/database/management/JanusGraphIndexWrapper.java index 817e97d7fd..c2c3bd9a10 100644 --- a/janusgraph-core/src/main/java/org/janusgraph/graphdb/database/management/JanusGraphIndexWrapper.java +++ b/janusgraph-core/src/main/java/org/janusgraph/graphdb/database/management/JanusGraphIndexWrapper.java @@ -79,6 +79,15 @@ public PropertyKey[] getFieldKeys() { return keys; } + @Override + public String[] getInlineFieldKeys() { + if (index.isMixedIndex()) { + return new String[0]; + } else { + return ((CompositeIndexType) index).getInlineFieldKeys(); + } + } + @Override public Parameter[] getParametersFor(PropertyKey key) { if (index.isCompositeIndex()) return new Parameter[0]; diff --git a/janusgraph-core/src/main/java/org/janusgraph/graphdb/database/management/ManagementSystem.java b/janusgraph-core/src/main/java/org/janusgraph/graphdb/database/management/ManagementSystem.java index 0e43eb9e39..0708429909 100644 --- a/janusgraph-core/src/main/java/org/janusgraph/graphdb/database/management/ManagementSystem.java +++ b/janusgraph-core/src/main/java/org/janusgraph/graphdb/database/management/ManagementSystem.java @@ -724,7 +724,32 @@ public void addIndexKey(final JanusGraphIndex index, final PropertyKey key, Para if (!key.isNew()) updateIndex(index, SchemaAction.REGISTER_INDEX); } - private JanusGraphIndex createCompositeIndex(String indexName, ElementCategory elementCategory, boolean unique, JanusGraphSchemaType constraint, PropertyKey... keys) { + @Override + public void addInlinePropertyKey(final JanusGraphIndex index, final PropertyKey key) { + Preconditions.checkArgument(index != null && key != null && index instanceof JanusGraphIndexWrapper + && !(key instanceof BaseKey), "Need to provide valid index and key"); + IndexType indexType = ((JanusGraphIndexWrapper) index).getBaseIndex(); + Preconditions.checkArgument(indexType instanceof CompositeIndexType, "Can only add keys to a composite index, not %s", index.name()); + Preconditions.checkArgument(indexType instanceof IndexTypeWrapper && key instanceof JanusGraphSchemaVertex + && ((IndexTypeWrapper) indexType).getSchemaBase() instanceof JanusGraphSchemaVertex); + + JanusGraphSchemaVertex indexVertex = (JanusGraphSchemaVertex) ((IndexTypeWrapper) indexType).getSchemaBase(); + + for (IndexField field : indexType.getFieldKeys()) + Preconditions.checkArgument(!field.getFieldKey().equals(key), "Key [%s] has already been added to index %s", key.name(), index.name()); + + addSchemaEdge(indexVertex, key, TypeDefinitionCategory.INDEX_INLINE_KEY, null); + updateSchemaVertex(indexVertex); + indexType.resetCache(); + + if (!indexVertex.isNew()) updatedTypes.add(indexVertex); + } + + private JanusGraphIndex createCompositeIndex(String indexName, ElementCategory elementCategory, + boolean unique, + JanusGraphSchemaType constraint, + Set inlineProps, + PropertyKey... keys) { checkIndexName(indexName); Preconditions.checkArgument(keys != null && keys.length > 0, "Need to provide keys to index [%s]", indexName); Preconditions.checkArgument(!unique || elementCategory == ElementCategory.VERTEX, "Unique indexes can only be created on vertices [%s]", indexName); @@ -756,6 +781,10 @@ private JanusGraphIndex createCompositeIndex(String indexName, ElementCategory e addSchemaEdge(indexVertex, keys[i], TypeDefinitionCategory.INDEX_FIELD, paras); } + for(PropertyKey propertyKey: inlineProps) { + addSchemaEdge(indexVertex, propertyKey, TypeDefinitionCategory.INDEX_INLINE_KEY, null); + } + Preconditions.checkArgument(constraint == null || (elementCategory.isValidConstraint(constraint) && constraint instanceof JanusGraphSchemaVertex)); if (constraint != null) { addSchemaEdge(indexVertex, (JanusGraphSchemaVertex) constraint, TypeDefinitionCategory.INDEX_SCHEMA_CONSTRAINT, null); @@ -779,6 +808,8 @@ private class IndexBuilder implements JanusGraphManagement.IndexBuilder { private JanusGraphSchemaType constraint = null; private final Map keys = new HashMap<>(); + private final Set inlinePropKeys = new HashSet<>(); + private IndexBuilder(String indexName, ElementCategory elementCategory) { this.indexName = indexName; this.elementCategory = elementCategory; @@ -798,6 +829,13 @@ public JanusGraphManagement.IndexBuilder addKey(PropertyKey key, Parameter... pa return this; } + @Override + public JanusGraphManagement.IndexBuilder addInlinePropertyKey(PropertyKey key) { + Preconditions.checkArgument(key != null && (key instanceof PropertyKeyVertex), "Key must be a user defined key: %s", key); + inlinePropKeys.add(key); + return this; + } + @Override public JanusGraphManagement.IndexBuilder indexOnly(JanusGraphSchemaType schemaType) { Preconditions.checkNotNull(schemaType); @@ -821,13 +859,14 @@ public JanusGraphIndex buildCompositeIndex() { Preconditions.checkArgument(entry.getValue() == null, "Cannot specify parameters for composite index: %s", entry.getKey()); keyArr[pos++] = entry.getKey(); } - return createCompositeIndex(indexName, elementCategory, unique, constraint, keyArr); + return createCompositeIndex(indexName, elementCategory, unique, constraint, inlinePropKeys, keyArr); } @Override public JanusGraphIndex buildMixedIndex(String backingIndex) { Preconditions.checkArgument(StringUtils.isNotBlank(backingIndex), "Need to specify backing index name"); Preconditions.checkArgument(!unique, "An external index cannot be unique"); + Preconditions.checkArgument(inlinePropKeys.isEmpty(), "An external index cannot contain inline properties"); JanusGraphIndex index = createMixedIndex(indexName, elementCategory, constraint, backingIndex); for (Map.Entry entry : keys.entrySet()) { diff --git a/janusgraph-core/src/main/java/org/janusgraph/graphdb/database/util/IndexRecordUtil.java b/janusgraph-core/src/main/java/org/janusgraph/graphdb/database/util/IndexRecordUtil.java index ced7bed9a1..6a37cd16da 100644 --- a/janusgraph-core/src/main/java/org/janusgraph/graphdb/database/util/IndexRecordUtil.java +++ b/janusgraph-core/src/main/java/org/janusgraph/graphdb/database/util/IndexRecordUtil.java @@ -18,6 +18,9 @@ import com.google.common.collect.ImmutableList; import com.google.common.collect.Iterators; import org.apache.commons.lang.StringUtils; +import org.apache.tinkerpop.gremlin.structure.Direction; +import org.apache.tinkerpop.gremlin.structure.VertexProperty; +import org.apache.tinkerpop.gremlin.util.iterator.IteratorUtils; import org.janusgraph.core.Cardinality; import org.janusgraph.core.JanusGraphElement; import org.janusgraph.core.JanusGraphRelation; @@ -30,8 +33,10 @@ import org.janusgraph.diskstorage.StaticBuffer; import org.janusgraph.diskstorage.indexing.IndexEntry; import org.janusgraph.diskstorage.indexing.StandardKeyInformation; +import org.janusgraph.diskstorage.keycolumnvalue.SliceQuery; import org.janusgraph.diskstorage.util.HashingUtil; import org.janusgraph.diskstorage.util.StaticArrayEntry; +import org.janusgraph.graphdb.database.EdgeSerializer; import org.janusgraph.graphdb.database.IndexRecordEntry; import org.janusgraph.graphdb.database.StandardJanusGraph; import org.janusgraph.graphdb.database.idhandling.IDHandler; @@ -47,6 +52,7 @@ import org.janusgraph.graphdb.internal.InternalRelation; import org.janusgraph.graphdb.internal.InternalRelationType; import org.janusgraph.graphdb.internal.InternalVertex; +import org.janusgraph.graphdb.olap.QueryContainer; import org.janusgraph.graphdb.query.vertex.VertexCentricQueryBuilder; import org.janusgraph.graphdb.relations.RelationIdentifier; import org.janusgraph.graphdb.transaction.StandardJanusGraphTx; @@ -56,13 +62,17 @@ import org.janusgraph.graphdb.types.MixedIndexType; import org.janusgraph.graphdb.types.ParameterIndexField; import org.janusgraph.graphdb.types.ParameterType; +import org.janusgraph.graphdb.types.TypeInspector; import org.janusgraph.util.IDUtils; import org.janusgraph.util.encoding.LongEncoding; import java.util.ArrayList; import java.util.Arrays; +import java.util.Collections; +import java.util.HashMap; import java.util.Iterator; import java.util.List; +import java.util.Map; import static org.janusgraph.util.encoding.LongEncoding.STRING_ENCODING_MARKER; @@ -166,9 +176,9 @@ public static StandardKeyInformation getKeyInformation(final ParameterIndexField return new StandardKeyInformation(field.getFieldKey(),field.getParameters()); } - public static IndexMutationType getUpdateType(InternalRelation relation) { + public static IndexMutationType getUpdateType(InternalRelation relation, boolean isInlined) { assert relation.isNew() || relation.isRemoved(); - return (relation.isNew()? IndexMutationType.ADD : IndexMutationType.DELETE); + return isInlined ? IndexMutationType.UPDATE : (relation.isNew() ? IndexMutationType.ADD : IndexMutationType.DELETE); } public static boolean indexAppliesTo(IndexType index, JanusGraphElement element) { @@ -275,10 +285,18 @@ public static void indexMatches(JanusGraphVertex vertex, IndexRecordEntry[] curr } - private static Entry getIndexEntry(CompositeIndexType index, IndexRecordEntry[] record, JanusGraphElement element, Serializer serializer) { - final DataOutput out = serializer.getDataOutput(1+8+8*record.length+4*8); + private static Entry getIndexEntry(CompositeIndexType index, IndexRecordEntry[] record, + JanusGraphElement element, + Serializer serializer, + TypeInspector typeInspector, + EdgeSerializer edgeSerializer) { + + List inlineProperties = getInlineProperties(element, index, typeInspector, edgeSerializer); + int inlinePropertiesSize = getInlinePropertiesSize(inlineProperties); + + final DataOutput out = serializer.getDataOutput(1 + 8 + 8 * record.length + 4 * 8 + inlinePropertiesSize); out.putByte(FIRST_INDEX_COLUMN_BYTE); - if (index.getCardinality()!=Cardinality.SINGLE) { + if (index.getCardinality() != Cardinality.SINGLE) { if (element instanceof JanusGraphVertex) { IDHandler.writeVertexId(out, element.id(), true); } else { @@ -286,18 +304,19 @@ private static Entry getIndexEntry(CompositeIndexType index, IndexRecordEntry[] assert ((JanusGraphRelation) element).longId() == ((RelationIdentifier) element.id()).getRelationId(); VariableLong.writePositive(out, ((JanusGraphRelation) element).longId()); } - if (index.getCardinality()!=Cardinality.SET) { + if (index.getCardinality() != Cardinality.SET) { for (final IndexRecordEntry re : record) { VariableLong.writePositive(out, re.getRelationId()); } } } - final int valuePosition=out.getPosition(); + final int valuePosition = out.getPosition(); if (element instanceof JanusGraphVertex) { IDHandler.writeVertexId(out, element.id(), true); + writeInlineProperties(inlineProperties, out); } else { assert element instanceof JanusGraphRelation; - final RelationIdentifier rid = (RelationIdentifier)element.id(); + final RelationIdentifier rid = (RelationIdentifier) element.id(); VariableLong.writePositive(out, rid.getRelationId()); IDHandler.writeVertexId(out, rid.getOutVertexId(), true); VariableLong.writePositive(out, rid.getTypeId()); @@ -305,7 +324,7 @@ private static Entry getIndexEntry(CompositeIndexType index, IndexRecordEntry[] IDHandler.writeVertexId(out, rid.getInVertexId(), true); } } - return new StaticArrayEntry(out.getStaticBuffer(),valuePosition); + return new StaticArrayEntry(out.getStaticBuffer(), valuePosition); } public static StaticBuffer getIndexKey(CompositeIndexType index, IndexRecordEntry[] record, Serializer serializer, boolean hashKeys, HashingUtil.HashLength hashLength) { @@ -339,14 +358,79 @@ public static long getIndexIdFromKey(StaticBuffer key, boolean hashKeys, Hashing } public static IndexUpdate getCompositeIndexUpdate(CompositeIndexType index, IndexMutationType indexMutationType, IndexRecordEntry[] record, - JanusGraphElement element, Serializer serializer, boolean hashKeys, HashingUtil.HashLength hashLength){ + JanusGraphElement element, + Serializer serializer, + TypeInspector typeInspector, + EdgeSerializer edgeSerializer, + boolean hashKeys, + HashingUtil.HashLength hashLength){ return new IndexUpdate<>(index, indexMutationType, getIndexKey(index, record, serializer, hashKeys, hashLength), - getIndexEntry(index, record, element, serializer), element); + getIndexEntry(index, record, element, serializer, typeInspector, edgeSerializer), element); } public static IndexUpdate getMixedIndexUpdate(JanusGraphElement element, PropertyKey key, Object value, MixedIndexType index, IndexMutationType updateType) { return new IndexUpdate<>(index, updateType, element2String(element), new IndexEntry(key2Field(index.getField(key)), value), element); } + + public static int getInlinePropertiesSize(List inlineProperties) { + return inlineProperties.size() * Integer.BYTES * 2 + inlineProperties.stream().mapToInt(StaticBuffer::length).sum(); + } + + public static void writeInlineProperties(List inlineProperties, DataOutput out) { + inlineProperties.forEach(entry -> { + out.putInt(entry.length()); + out.putInt(entry.getValuePosition()); + out.putBytes(entry); + }); + } + + public static Iterable readInlineProperties(ReadBuffer readBuffer) { + + return () -> new Iterator() { + @Override + public boolean hasNext() { + return readBuffer.hasRemaining(); + } + + @Override + public Entry next() { + int entryDataSize = readBuffer.getInt(); + int valuePos = readBuffer.getInt(); + byte[] entryBytes = readBuffer.getBytes(entryDataSize); + return new StaticArrayEntry(entryBytes, valuePos); + } + }; + } + + public static List getInlineProperties(JanusGraphElement element, + CompositeIndexType index, + TypeInspector typeInspector, + EdgeSerializer edgeSerializer) { + if (element instanceof JanusGraphVertex && index.getInlineFieldKeys().length != 0 && !element.isRemoved()) { + Iterator> props = ((JanusGraphVertex) element).properties(index.getInlineFieldKeys()); + return IteratorUtils.list(IteratorUtils.map(props, + prop -> edgeSerializer.writeRelation((InternalRelation) prop, 0, typeInspector))); + } else { + return Collections.emptyList(); + } + } + + public static Map getInlinePropertiesQueries(CompositeIndexType index, StandardJanusGraphTx tx) { + if (index.getInlineFieldKeys().length == 0) { + return Collections.emptyMap(); + } else { + + Map result = new HashMap<>(index.getInlineFieldKeys().length); + for(String inlineKey: index.getInlineFieldKeys()) { + QueryContainer qc = new QueryContainer(tx); + qc.addQuery().direction(Direction.OUT).keys(inlineKey).properties(); + List sliceQueries = qc.getSliceQueries(); + assert sliceQueries.size() == 1; + result.put(inlineKey, sliceQueries.get(0)); + } + return result; + } + } } diff --git a/janusgraph-core/src/main/java/org/janusgraph/graphdb/database/util/StaleIndexRecordUtil.java b/janusgraph-core/src/main/java/org/janusgraph/graphdb/database/util/StaleIndexRecordUtil.java index 00a7fd6eab..0a504b0170 100644 --- a/janusgraph-core/src/main/java/org/janusgraph/graphdb/database/util/StaleIndexRecordUtil.java +++ b/janusgraph-core/src/main/java/org/janusgraph/graphdb/database/util/StaleIndexRecordUtil.java @@ -24,6 +24,7 @@ import org.janusgraph.diskstorage.StaticBuffer; import org.janusgraph.diskstorage.indexing.IndexTransaction; import org.janusgraph.diskstorage.util.HashingUtil; +import org.janusgraph.graphdb.database.EdgeSerializer; import org.janusgraph.graphdb.database.IndexRecordEntry; import org.janusgraph.graphdb.database.StandardJanusGraph; import org.janusgraph.graphdb.database.index.IndexMutationType; @@ -333,6 +334,7 @@ private static void forceRemoveElementFromCompositeIndex(JanusGraphElement eleme verifyIndexIsComposite(index); Serializer serializer = graph.getDataSerializer(); + EdgeSerializer edgeSerializer = graph.getEdgeSerializer(); boolean hashKeys = graph.getIndexSerializer().isHashKeys(); HashingUtil.HashLength hashLength = graph.getIndexSerializer().getHashLength(); @@ -340,19 +342,21 @@ private static void forceRemoveElementFromCompositeIndex(JanusGraphElement eleme CompositeIndexType compositeIndexType = (CompositeIndexType) indexSchemaVertex.asIndexType(); + StandardJanusGraphTx tx = (StandardJanusGraphTx) graph.newTransaction(); + BackendTransaction transaction = tx.getTxHandle(); + IndexUpdate update = IndexRecordUtil.getCompositeIndexUpdate( compositeIndexType, IndexMutationType.DELETE, indexRecord, elementToRemoveFromIndex, serializer, + tx, + edgeSerializer, hashKeys, hashLength ); - StandardJanusGraphTx tx = (StandardJanusGraphTx) graph.newTransaction(); - BackendTransaction transaction = tx.getTxHandle(); - try{ transaction.mutateIndex(update.getKey(), Collections.emptyList(), Collections.singletonList(update.getEntry())); } finally { diff --git a/janusgraph-core/src/main/java/org/janusgraph/graphdb/internal/InternalRelationType.java b/janusgraph-core/src/main/java/org/janusgraph/graphdb/internal/InternalRelationType.java index cec966d922..cf27588ead 100644 --- a/janusgraph-core/src/main/java/org/janusgraph/graphdb/internal/InternalRelationType.java +++ b/janusgraph-core/src/main/java/org/janusgraph/graphdb/internal/InternalRelationType.java @@ -20,6 +20,7 @@ import org.janusgraph.core.schema.ConsistencyModifier; import org.janusgraph.core.schema.SchemaStatus; import org.janusgraph.graphdb.types.IndexType; +import org.janusgraph.graphdb.types.indextype.IndexReferenceType; /** * Internal Type interface adding methods that should only be used by JanusGraph @@ -51,4 +52,6 @@ public interface InternalRelationType extends RelationType, InternalVertex { SchemaStatus getStatus(); Iterable getKeyIndexes(); + + Iterable getKeyIndexesReferences(); } diff --git a/janusgraph-core/src/main/java/org/janusgraph/graphdb/olap/job/IndexRepairJob.java b/janusgraph-core/src/main/java/org/janusgraph/graphdb/olap/job/IndexRepairJob.java index e71a9f3d79..43144a066c 100644 --- a/janusgraph-core/src/main/java/org/janusgraph/graphdb/olap/job/IndexRepairJob.java +++ b/janusgraph-core/src/main/java/org/janusgraph/graphdb/olap/job/IndexRepairJob.java @@ -200,7 +200,7 @@ public void process(JanusGraphVertex vertex, ScanMetrics metrics) { while (elements.hasNext()) { JanusGraphElement element = elements.next(); Set> updates = - indexSerializer.reindexElement(element, (CompositeIndexType) indexType); + indexSerializer.reindexElement(element, (CompositeIndexType) indexType, writeTx); for (IndexUpdate update : updates) { log.debug("Mutating index {}: {}", indexType, update.getEntry()); mutator.mutateIndex(update.getKey(), new ArrayList(1){{add(update.getEntry());}}, KCVSCache.NO_DELETIONS); diff --git a/janusgraph-core/src/main/java/org/janusgraph/graphdb/query/graph/GraphCentricQueryBuilder.java b/janusgraph-core/src/main/java/org/janusgraph/graphdb/query/graph/GraphCentricQueryBuilder.java index bfc51f394b..b54953483a 100644 --- a/janusgraph-core/src/main/java/org/janusgraph/graphdb/query/graph/GraphCentricQueryBuilder.java +++ b/janusgraph-core/src/main/java/org/janusgraph/graphdb/query/graph/GraphCentricQueryBuilder.java @@ -296,13 +296,15 @@ public GraphCentricQuery constructQueryWithoutProfile(final ElementCategory resu BackendQueryHolder query; if (!coveredClauses.isEmpty()) { int indexLimit; + int uncoveredAndConditions = 0; if (useSmartLimit) { indexLimit = limit == Query.NO_LIMIT ? DEFAULT_NO_LIMIT : Math.min(MAX_BASE_LIMIT, limit); + uncoveredAndConditions = coveredClauses.size(); } else { indexLimit = limit == Query.NO_LIMIT ? hardMaxLimit : limit; } indexLimit = Math.min(hardMaxLimit, - QueryUtil.adjustLimitForTxModifications(tx, conditions.numChildren() - coveredClauses.size(), indexLimit)); + QueryUtil.adjustLimitForTxModifications(tx, conditions.numChildren() - uncoveredAndConditions, indexLimit)); query = new BackendQueryHolder<>(selectedIndex.getQuery().updateLimit(indexLimit), coveredClauses.size() == conditions.numChildren() || coveredClauses.contains(conditions), selectedIndex.isSorted()); } else { diff --git a/janusgraph-core/src/main/java/org/janusgraph/graphdb/query/vertex/VertexWithInlineProps.java b/janusgraph-core/src/main/java/org/janusgraph/graphdb/query/vertex/VertexWithInlineProps.java new file mode 100644 index 0000000000..ec0baac01f --- /dev/null +++ b/janusgraph-core/src/main/java/org/janusgraph/graphdb/query/vertex/VertexWithInlineProps.java @@ -0,0 +1,89 @@ +// Copyright 2024 JanusGraph Authors +// +// Licensed under the Apache License, Version 2.0 (the "License"); +// you may not use this file except in compliance with the License. +// You may obtain a copy of the License at +// +// http://www.apache.org/licenses/LICENSE-2.0 +// +// Unless required by applicable law or agreed to in writing, software +// distributed under the License is distributed on an "AS IS" BASIS, +// WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. +// See the License for the specific language governing permissions and +// limitations under the License. + +package org.janusgraph.graphdb.query.vertex; + +import org.janusgraph.diskstorage.Entry; +import org.janusgraph.diskstorage.EntryList; +import org.janusgraph.diskstorage.keycolumnvalue.SliceQuery; +import org.janusgraph.diskstorage.util.EntryArrayList; +import org.janusgraph.graphdb.internal.InternalRelationType; +import org.janusgraph.graphdb.transaction.StandardJanusGraphTx; +import org.slf4j.Logger; +import org.slf4j.LoggerFactory; + +import java.util.Collections; +import java.util.HashMap; +import java.util.Map; + +public class VertexWithInlineProps { + private final Object vertexId; + private final Map inlineProperties; + + private static final Logger log = LoggerFactory.getLogger(VertexWithInlineProps.class); + + public VertexWithInlineProps(Object vertexId, EntryList inlineProperties, Map inlineQueries, StandardJanusGraphTx tx) { + this.vertexId = vertexId; + this.inlineProperties = loadInlineProperties(inlineProperties, inlineQueries, tx); + } + + public Object getVertexId() { + return vertexId; + } + + public Map getInlineProperties() { + return inlineProperties; + } + + private Map loadInlineProperties(EntryList inlineProperties, + Map inlineQueries, + StandardJanusGraphTx tx) { + if (inlineProperties.isEmpty()) { + return Collections.emptyMap(); + } else { + Map result = new HashMap<>(); + for (Entry dataEntry : inlineProperties) { + long typeId = tx.getEdgeSerializer().parseTypeId(dataEntry); + InternalRelationType type = tx.getOrLoadRelationTypeById(typeId); + assert type.isPropertyKey(); + + SliceQuery sq = inlineQueries.get(type.name()); + if(sq != null) { + if (result.containsKey(sq)) { + result.get(sq).add(dataEntry); + } else { + EntryList entryList = new EntryArrayList(); + entryList.add(dataEntry); + result.put(sq, entryList); + } + } else { + log.error("Missing key=" + type.name() + " in inlineQueries. Check index definition."); + } + } + return result; + } + } + + @Override + public int hashCode() { + return vertexId.hashCode(); + } + + @Override + public boolean equals(Object oth) { + if (this == oth) return true; + else if (oth == null || !getClass().isInstance(oth)) return false; + return vertexId == (((VertexWithInlineProps) oth).vertexId); + } +} diff --git a/janusgraph-core/src/main/java/org/janusgraph/graphdb/transaction/StandardJanusGraphTx.java b/janusgraph-core/src/main/java/org/janusgraph/graphdb/transaction/StandardJanusGraphTx.java index bc0cc32f0d..712b44c8ed 100644 --- a/janusgraph-core/src/main/java/org/janusgraph/graphdb/transaction/StandardJanusGraphTx.java +++ b/janusgraph-core/src/main/java/org/janusgraph/graphdb/transaction/StandardJanusGraphTx.java @@ -88,6 +88,7 @@ import org.janusgraph.graphdb.query.vertex.MultiVertexCentricQueryBuilder; import org.janusgraph.graphdb.query.vertex.VertexCentricQuery; import org.janusgraph.graphdb.query.vertex.VertexCentricQueryBuilder; +import org.janusgraph.graphdb.query.vertex.VertexWithInlineProps; import org.janusgraph.graphdb.relations.RelationComparator; import org.janusgraph.graphdb.relations.RelationIdentifier; import org.janusgraph.graphdb.relations.RelationIdentifierUtils; @@ -1125,7 +1126,7 @@ public boolean containsEdgeLabel(String name) { return type!=null && type.isEdgeLabel(); } - // this is critical path we can't allow anything heavier then assertion in here + // this is critical path we can't allow anything heavier than assertion in here @Override public RelationType getExistingRelationType(long typeId) { assert idInspector.isRelationTypeId(typeId); @@ -1502,7 +1503,7 @@ public Iterator execute(final GraphCentricQuery query, final final JointIndexQuery.Subquery adjustedQuery = subquery.updateLimit(limit); try { return indexCache.get(adjustedQuery, - () -> QueryProfiler.profile(subquery.getProfiler(), adjustedQuery, q -> indexSerializer.query(q, txHandle).collect(Collectors.toList()))); + () -> QueryProfiler.profile(subquery.getProfiler(), adjustedQuery, q -> indexSerializer.query(q, txHandle, StandardJanusGraphTx.this).collect(Collectors.toList()))); } catch (Exception e) { throw new JanusGraphException("Could not call index", e); } @@ -1510,7 +1511,7 @@ public Iterator execute(final GraphCentricQuery query, final } // Constructs an iterator which lazily streams results from 1st index, and filters by looking up in the intersection of results from all other indices (if any) // NOTE NO_LIMIT is passed to processIntersectingRetrievals to prevent incomplete intersections, which could lead to missed results - iterator = new SubqueryIterator(indexQuery.getQuery(0), indexSerializer, txHandle, indexCache, indexQuery.getLimit(), getConversionFunction(query.getResultType()), + iterator = new SubqueryIterator(indexQuery.getQuery(0), indexSerializer, txHandle, StandardJanusGraphTx.this, indexCache, indexQuery.getLimit(), getConversionFunction(query.getResultType()), retrievals.isEmpty() ? null: QueryUtil.processIntersectingRetrievals(retrievals, Query.NO_LIMIT)); } else { if (config.hasForceIndexUsage()) throw new JanusGraphException("Could not find a suitable index to answer graph query and graph scans are disabled: " + query); @@ -1558,7 +1559,13 @@ public Iterator execute(final GraphCentricQuery query, final private final Function vertexIDConversionFct = id -> { Preconditions.checkNotNull(id); - return getInternalVertex(id); + Preconditions.checkArgument(id instanceof VertexWithInlineProps); + VertexWithInlineProps v = (VertexWithInlineProps) id; + InternalVertex vertex = getInternalVertex(v.getVertexId()); + if (vertex instanceof CacheVertex) { + v.getInlineProperties().forEach((sq, entryList) -> ((CacheVertex) vertex).addToQueryCache(sq, entryList)); + } + return vertex; }; private final Function edgeIDConversionFct = id -> { diff --git a/janusgraph-core/src/main/java/org/janusgraph/graphdb/types/CompositeIndexType.java b/janusgraph-core/src/main/java/org/janusgraph/graphdb/types/CompositeIndexType.java index 30e980d9ed..cf9951f071 100644 --- a/janusgraph-core/src/main/java/org/janusgraph/graphdb/types/CompositeIndexType.java +++ b/janusgraph-core/src/main/java/org/janusgraph/graphdb/types/CompositeIndexType.java @@ -23,6 +23,8 @@ */ public interface CompositeIndexType extends IndexType { + static final String[] EMPTY_INLINE_PROPS = new String[0]; + /** * @deprecated use longId() * @return return index id @@ -32,6 +34,8 @@ public interface CompositeIndexType extends IndexType { IndexField[] getFieldKeys(); + String[] getInlineFieldKeys(); + SchemaStatus getStatus(); /* diff --git a/janusgraph-core/src/main/java/org/janusgraph/graphdb/types/TypeDefinitionCategory.java b/janusgraph-core/src/main/java/org/janusgraph/graphdb/types/TypeDefinitionCategory.java index cb1c97ea1a..8e7b2d0f6f 100644 --- a/janusgraph-core/src/main/java/org/janusgraph/graphdb/types/TypeDefinitionCategory.java +++ b/janusgraph-core/src/main/java/org/janusgraph/graphdb/types/TypeDefinitionCategory.java @@ -75,7 +75,8 @@ public enum TypeDefinitionCategory { INDEX_SCHEMA_CONSTRAINT(), PROPERTY_KEY_EDGE(), CONNECTION_EDGE(RelationCategory.EDGE, String.class), - UPDATE_CONNECTION_EDGE(); + UPDATE_CONNECTION_EDGE(), + INDEX_INLINE_KEY(); public static final Set PROPERTYKEY_DEFINITION_CATEGORIES = Collections.unmodifiableSet(new HashSet<>(Arrays.asList(STATUS, INVISIBLE, SORT_KEY, SORT_ORDER, SIGNATURE, MULTIPLICITY, DATATYPE))); public static final Set EDGELABEL_DEFINITION_CATEGORIES = Collections.unmodifiableSet(new HashSet<>(Arrays.asList(STATUS, INVISIBLE, SORT_KEY, SORT_ORDER, SIGNATURE, MULTIPLICITY, UNIDIRECTIONAL))); diff --git a/janusgraph-core/src/main/java/org/janusgraph/graphdb/types/indextype/CompositeIndexTypeWrapper.java b/janusgraph-core/src/main/java/org/janusgraph/graphdb/types/indextype/CompositeIndexTypeWrapper.java index 4d21d5ff73..8294fda446 100644 --- a/janusgraph-core/src/main/java/org/janusgraph/graphdb/types/indextype/CompositeIndexTypeWrapper.java +++ b/janusgraph-core/src/main/java/org/janusgraph/graphdb/types/indextype/CompositeIndexTypeWrapper.java @@ -15,6 +15,7 @@ package org.janusgraph.graphdb.types.indextype; import com.google.common.base.Preconditions; +import com.google.common.collect.Iterables; import org.apache.tinkerpop.gremlin.structure.Direction; import org.janusgraph.core.Cardinality; import org.janusgraph.core.PropertyKey; @@ -37,6 +38,8 @@ public class CompositeIndexTypeWrapper extends IndexTypeWrapper implements Compo private IndexField[] fields = null; + private String[] inlineKeys = null; + public CompositeIndexTypeWrapper(SchemaSource base) { super(base); } @@ -82,10 +85,29 @@ public IndexField[] getFieldKeys() { return result; } + @Override + public String[] getInlineFieldKeys() { + String[] result = inlineKeys; + if (result == null) { + List entries = base.getRelated(TypeDefinitionCategory.INDEX_INLINE_KEY, Direction.OUT); + int numFields = entries.size(); + result = new String[numFields]; + int pos = 0; + for (SchemaSource.Entry entry : entries) { + assert entry.getSchemaType() instanceof PropertyKey; + result[pos] = ((PropertyKey) entry.getSchemaType()).name(); + pos++; + } + inlineKeys = result; + } + return result; + } + @Override public void resetCache() { super.resetCache(); fields = null; + inlineKeys = null; } @Override diff --git a/janusgraph-core/src/main/java/org/janusgraph/graphdb/types/indextype/IndexReferenceType.java b/janusgraph-core/src/main/java/org/janusgraph/graphdb/types/indextype/IndexReferenceType.java new file mode 100644 index 0000000000..696e795e9e --- /dev/null +++ b/janusgraph-core/src/main/java/org/janusgraph/graphdb/types/indextype/IndexReferenceType.java @@ -0,0 +1,37 @@ +// Copyright 2024 JanusGraph Authors +// +// Licensed under the Apache License, Version 2.0 (the "License"); +// you may not use this file except in compliance with the License. +// You may obtain a copy of the License at +// +// http://www.apache.org/licenses/LICENSE-2.0 +// +// Unless required by applicable law or agreed to in writing, software +// distributed under the License is distributed on an "AS IS" BASIS, +// WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. +// See the License for the specific language governing permissions and +// limitations under the License. + +package org.janusgraph.graphdb.types.indextype; + +import org.janusgraph.graphdb.types.IndexType; + +public class IndexReferenceType { + + private final boolean isInlined; + + private final IndexType indexType; + + public IndexReferenceType(boolean isInlined, IndexType indexType) { + this.isInlined = isInlined; + this.indexType = indexType; + } + + public boolean isInlined() { + return isInlined; + } + + public IndexType getIndexType() { + return indexType; + } +} diff --git a/janusgraph-core/src/main/java/org/janusgraph/graphdb/types/system/BaseKey.java b/janusgraph-core/src/main/java/org/janusgraph/graphdb/types/system/BaseKey.java index 16198ce659..82807fb93e 100644 --- a/janusgraph-core/src/main/java/org/janusgraph/graphdb/types/system/BaseKey.java +++ b/janusgraph-core/src/main/java/org/janusgraph/graphdb/types/system/BaseKey.java @@ -15,6 +15,7 @@ package org.janusgraph.graphdb.types.system; import com.google.common.base.Preconditions; +import com.google.common.collect.Iterables; import org.apache.tinkerpop.gremlin.structure.Direction; import org.janusgraph.core.Cardinality; import org.janusgraph.core.Multiplicity; @@ -29,6 +30,7 @@ import org.janusgraph.graphdb.types.IndexField; import org.janusgraph.graphdb.types.IndexType; import org.janusgraph.graphdb.types.TypeDefinitionDescription; +import org.janusgraph.graphdb.types.indextype.IndexReferenceType; import java.util.Collections; @@ -105,6 +107,11 @@ public Iterable getKeyIndexes() { return Collections.singletonList(indexDef); } + @Override + public Iterable getKeyIndexesReferences() { + return Iterables.transform(getKeyIndexes(), indexType -> new IndexReferenceType(false, indexType)); + } + private final CompositeIndexType indexDef = new CompositeIndexType() { private final IndexField[] fields = {IndexField.of(BaseKey.this)}; @@ -130,6 +137,11 @@ public IndexField[] getFieldKeys() { return fields; } + @Override + public String[] getInlineFieldKeys() { + return CompositeIndexType.EMPTY_INLINE_PROPS; + } + @Override public IndexField getField(PropertyKey key) { if (key.equals(BaseKey.this)) return fields[0]; diff --git a/janusgraph-core/src/main/java/org/janusgraph/graphdb/types/system/EmptyRelationType.java b/janusgraph-core/src/main/java/org/janusgraph/graphdb/types/system/EmptyRelationType.java index 2e2a30b36f..942121420a 100644 --- a/janusgraph-core/src/main/java/org/janusgraph/graphdb/types/system/EmptyRelationType.java +++ b/janusgraph-core/src/main/java/org/janusgraph/graphdb/types/system/EmptyRelationType.java @@ -18,6 +18,7 @@ import org.janusgraph.graphdb.internal.InternalRelationType; import org.janusgraph.graphdb.internal.Order; import org.janusgraph.graphdb.types.IndexType; +import org.janusgraph.graphdb.types.indextype.IndexReferenceType; import java.util.Collections; @@ -66,6 +67,11 @@ public Iterable getKeyIndexes() { return Collections.EMPTY_LIST; } + @Override + public Iterable getKeyIndexesReferences(){ + return Collections.EMPTY_LIST; + } + public Integer getTTL() { return 0; } diff --git a/janusgraph-core/src/main/java/org/janusgraph/graphdb/types/vertices/RelationTypeVertex.java b/janusgraph-core/src/main/java/org/janusgraph/graphdb/types/vertices/RelationTypeVertex.java index 5cec503967..37c7937d0d 100644 --- a/janusgraph-core/src/main/java/org/janusgraph/graphdb/types/vertices/RelationTypeVertex.java +++ b/janusgraph-core/src/main/java/org/janusgraph/graphdb/types/vertices/RelationTypeVertex.java @@ -22,12 +22,17 @@ import org.janusgraph.graphdb.internal.Order; import org.janusgraph.graphdb.transaction.StandardJanusGraphTx; import org.janusgraph.graphdb.types.IndexType; +import org.janusgraph.graphdb.types.SchemaSource; import org.janusgraph.graphdb.types.TypeDefinitionCategory; import org.janusgraph.graphdb.types.TypeUtil; +import org.janusgraph.graphdb.types.indextype.IndexReferenceType; import org.janusgraph.graphdb.util.CollectionsUtil; +import java.util.ArrayList; import java.util.Collections; +import java.util.HashMap; import java.util.List; +import java.util.Map; /** * @author Matthias Broecheler (me@matthiasb.com) @@ -38,6 +43,8 @@ public abstract class RelationTypeVertex extends JanusGraphSchemaVertex implemen private Integer ttl = null; private List indexes = null; + private List indexesReferences = null; + public RelationTypeVertex(StandardJanusGraphTx tx, Object id, byte lifecycle) { super(tx, id, lifecycle); } @@ -105,22 +112,56 @@ public Iterable getRelationIndexes() { @Override public Iterable getKeyIndexes() { List result = indexes; - if (result==null) { - result = Collections.unmodifiableList( - CollectionsUtil.toArrayList( - getRelated(TypeDefinitionCategory.INDEX_FIELD,Direction.IN), - entry -> entry.getSchemaType().asIndexType() - ) - ); - indexes=result; + if (result == null) { + result = getIndexes(); + indexes = result; + } + return result; + } + + @Override + public Iterable getKeyIndexesReferences() { + List result = indexesReferences; + if (result == null) { + result = getIndexesReferences(); + indexesReferences = result; } - assert result!=null; return result; } + private List getIndexes() { + return Collections.unmodifiableList( + CollectionsUtil.toArrayList( + getRelated(TypeDefinitionCategory.INDEX_FIELD, Direction.IN), + entry -> entry.getSchemaType().asIndexType() + ) + ); + } + + private List getIndexesReferences() { + Map relatedIndexes = new HashMap<>(); + + for (Entry entry : getRelated(TypeDefinitionCategory.INDEX_FIELD, Direction.IN)) { + SchemaSource index = entry.getSchemaType(); + IndexReferenceType item = new IndexReferenceType(false, index.asIndexType()); + relatedIndexes.put(index.name(), item); + } + + for (Entry entry : getRelated(TypeDefinitionCategory.INDEX_INLINE_KEY, Direction.IN)) { + SchemaSource index = entry.getSchemaType(); + if (!relatedIndexes.containsKey(index.name())) { + IndexReferenceType item = new IndexReferenceType(true, index.asIndexType()); + relatedIndexes.put(index.name(), item); + } + } + + return new ArrayList<>(relatedIndexes.values()); + } + @Override public void resetCache() { super.resetCache(); - indexes=null; + indexes = null; + indexesReferences = null; } } diff --git a/janusgraph-core/src/main/java/org/janusgraph/graphdb/util/SubqueryIterator.java b/janusgraph-core/src/main/java/org/janusgraph/graphdb/util/SubqueryIterator.java index cb3b4a6be4..ea27a4f274 100644 --- a/janusgraph-core/src/main/java/org/janusgraph/graphdb/util/SubqueryIterator.java +++ b/janusgraph-core/src/main/java/org/janusgraph/graphdb/util/SubqueryIterator.java @@ -23,6 +23,7 @@ import org.janusgraph.graphdb.database.IndexSerializer; import org.janusgraph.graphdb.query.graph.JointIndexQuery; import org.janusgraph.graphdb.query.profile.QueryProfiler; +import org.janusgraph.graphdb.transaction.StandardJanusGraphTx; import org.janusgraph.graphdb.transaction.subquerycache.SubqueryCache; import org.slf4j.Logger; import org.slf4j.LoggerFactory; @@ -51,7 +52,9 @@ public class SubqueryIterator extends CloseableAbstractIterator function, List otherResults) { this.subQuery = subQuery; @@ -65,7 +68,7 @@ public SubqueryIterator(JointIndexQuery.Subquery subQuery, IndexSerializer index currentIds = new ArrayList<>(); profiler = QueryProfiler.startProfile(subQuery.getProfiler(), subQuery); isTimerRunning = true; - stream = indexSerializer.query(subQuery, tx).peek(r -> currentIds.add(r)); + stream = indexSerializer.query(subQuery, backendTx, tx).peek(r -> currentIds.add(r)); } catch (final Exception e) { throw new JanusGraphException("Could not call index", e); } diff --git a/janusgraph-core/src/main/java/org/janusgraph/graphdb/vertices/CacheVertex.java b/janusgraph-core/src/main/java/org/janusgraph/graphdb/vertices/CacheVertex.java index 46013396f5..8c46271784 100644 --- a/janusgraph-core/src/main/java/org/janusgraph/graphdb/vertices/CacheVertex.java +++ b/janusgraph-core/src/main/java/org/janusgraph/graphdb/vertices/CacheVertex.java @@ -48,7 +48,7 @@ public EntryList getFromCache(final SliceQuery query) { return queryCache.get(query); } - protected void addToQueryCache(final SliceQuery query, final EntryList entries) { + public void addToQueryCache(final SliceQuery query, final EntryList entries) { synchronized (queryCache) { //TODO: become smarter about what to cache and when (e.g. memory pressure) queryCache.put(query, entries); diff --git a/janusgraph-test/src/test/java/org/janusgraph/graphdb/database/IndexSerializerTest.java b/janusgraph-test/src/test/java/org/janusgraph/graphdb/database/IndexSerializerTest.java index 97bdb20c09..1f682b879b 100644 --- a/janusgraph-test/src/test/java/org/janusgraph/graphdb/database/IndexSerializerTest.java +++ b/janusgraph-test/src/test/java/org/janusgraph/graphdb/database/IndexSerializerTest.java @@ -52,9 +52,10 @@ public class IndexSerializerTest { public void testReindexElementNotAppliesTo() { Configuration config = mock(Configuration.class); Serializer serializer = mock(Serializer.class); + EdgeSerializer edgeSerializer = mock(EdgeSerializer.class); Map indexes = new HashMap<>(); - IndexSerializer mockSerializer = new IndexSerializer(config, serializer, indexes, true); + IndexSerializer mockSerializer = new IndexSerializer(config, edgeSerializer, serializer, indexes, true); JanusGraphElement nonIndexableElement = mock(JanusGraphElement.class); MixedIndexType mit = mock(MixedIndexType.class); doReturn(ElementCategory.VERTEX).when(mit).getElement(); @@ -92,8 +93,9 @@ public void testReindexElementAppliesToNoEntries() { private IndexSerializer mockSerializer() { Configuration config = mock(Configuration.class); Serializer serializer = mock(Serializer.class); + EdgeSerializer edgeSerializer = mock(EdgeSerializer.class); Map indexes = new HashMap<>(); - return spy(new IndexSerializer(config, serializer, indexes, true)); + return spy(new IndexSerializer(config, edgeSerializer, serializer, indexes, true)); } private JanusGraphElement mockIndexAppliesTo(MixedIndexType mit, boolean indexable) {