You are viewing a plain text version of this content. The canonical link for it is here.
Posted to notifications@ignite.apache.org by GitBox <gi...@apache.org> on 2022/08/05 08:13:34 UTC

[GitHub] [ignite-3] ibessonov commented on a diff in pull request #976: IGNITE-17085 Support persistent case for page-memory-based MV storage

ibessonov commented on code in PR #976:
URL: https://github.com/apache/ignite-3/pull/976#discussion_r938553216


##########
modules/storage-page-memory/src/main/java/org/apache/ignite/internal/storage/pagememory/AbstractPageMemoryPartitionStorage.java:
##########
@@ -0,0 +1,487 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements.  See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You 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.apache.ignite.internal.storage.pagememory;
+
+import static org.apache.ignite.internal.pagememory.PageIdAllocator.MAX_PARTITION_ID;
+
+import java.nio.ByteBuffer;
+import java.nio.file.Path;
+import java.util.ArrayList;
+import java.util.Collection;
+import java.util.Iterator;
+import java.util.List;
+import java.util.NoSuchElementException;
+import java.util.concurrent.CompletableFuture;
+import java.util.function.Predicate;
+import org.apache.ignite.internal.pagememory.tree.BplusTree;
+import org.apache.ignite.internal.pagememory.tree.IgniteTree;
+import org.apache.ignite.internal.storage.DataRow;
+import org.apache.ignite.internal.storage.InvokeClosure;
+import org.apache.ignite.internal.storage.OperationType;
+import org.apache.ignite.internal.storage.PartitionStorage;
+import org.apache.ignite.internal.storage.SearchRow;
+import org.apache.ignite.internal.storage.StorageException;
+import org.apache.ignite.internal.storage.StorageUtils;
+import org.apache.ignite.internal.util.Cursor;
+import org.apache.ignite.internal.util.IgniteCursor;
+import org.apache.ignite.lang.IgniteInternalCheckedException;
+import org.apache.ignite.lang.IgniteInternalException;
+import org.jetbrains.annotations.Nullable;
+
+/**
+ * Abstract implementation of {@link PartitionStorage} based on a {@link BplusTree}.
+ */
+// TODO: IGNITE-16644 Support snapshots.
+abstract class AbstractPageMemoryPartitionStorage implements PartitionStorage {

Review Comment:
   I don't get this class. PartitionStorage is deprecated, what's the point? Can you please explain?



##########
modules/storage-page-memory/src/main/java/org/apache/ignite/internal/storage/pagememory/PersistentPageMemoryTableStorage.java:
##########
@@ -164,8 +170,113 @@ public void destroy() throws StorageException {
 
     /** {@inheritDoc} */
     @Override
-    public PageMemoryMvPartitionStorage createMvPartitionStorage(int partitionId) {
-        throw new UnsupportedOperationException("Not supported yet");
+    public PersistentPageMemoryMvPartitionStorage createMvPartitionStorage(int partitionId) {

Review Comment:
   Do we need an old method for creating partition? It can just throw UnsupportedOperationException.
   
   Why did you rename partId to partitionId in method that has nothing to do with your code btw? Why was it necessary?



##########
modules/storage-page-memory/src/main/java/org/apache/ignite/internal/storage/pagememory/VolatilePageMemoryDataRegion.java:
##########
@@ -104,60 +103,43 @@ public void start() {
     }
 
     private TableFreeList createTableFreeList(PageMemory pageMemory) throws IgniteInternalCheckedException {
-        long metaPageId = pageMemory.allocatePage(FREE_LIST_GROUP_ID, FREE_LIST_PARTITION_ID, FLAG_AUX);
-
         return new TableFreeList(
                 FREE_LIST_GROUP_ID,
                 FREE_LIST_PARTITION_ID,
                 pageMemory,
                 PageLockListenerNoOp.INSTANCE,
-                metaPageId,
+                pageMemory.allocatePage(FREE_LIST_GROUP_ID, FREE_LIST_PARTITION_ID, FLAG_AUX),

Review Comment:
   Why did you inline this? Old code was clean enough, and new code became more complicated. I don't understand the necessity.



##########
modules/storage-page-memory/src/main/java/org/apache/ignite/internal/storage/pagememory/mv/RowVersionFreeList.java:
##########
@@ -83,7 +80,7 @@ public RowVersionFreeList(
                 LOG,
                 metaPageId,
                 initNew,
-                pageListCacheLimit,
+                null,

Review Comment:
   If this thing doesn't exist anymore then why do we have this "null" argument?



##########
modules/storage-page-memory/src/main/java/org/apache/ignite/internal/storage/pagememory/VolatilePageMemoryDataRegion.java:
##########
@@ -104,60 +103,43 @@ public void start() {
     }
 
     private TableFreeList createTableFreeList(PageMemory pageMemory) throws IgniteInternalCheckedException {
-        long metaPageId = pageMemory.allocatePage(FREE_LIST_GROUP_ID, FREE_LIST_PARTITION_ID, FLAG_AUX);
-
         return new TableFreeList(
                 FREE_LIST_GROUP_ID,
                 FREE_LIST_PARTITION_ID,
                 pageMemory,
                 PageLockListenerNoOp.INSTANCE,
-                metaPageId,
+                pageMemory.allocatePage(FREE_LIST_GROUP_ID, FREE_LIST_PARTITION_ID, FLAG_AUX),
                 true,
-                null,
                 PageEvictionTrackerNoOp.INSTANCE,
                 IoStatisticsHolderNoOp.INSTANCE
         );
     }
 
-    private static VersionChainFreeList createVersionChainFreeList(
-            PageMemory pageMemory,
-            ReuseList reuseList
-    ) throws IgniteInternalCheckedException {
-        long metaPageId = pageMemory.allocatePage(FREE_LIST_GROUP_ID, FREE_LIST_PARTITION_ID, FLAG_AUX);
-
+    private static VersionChainFreeList createVersionChainFreeList(PageMemory pageMemory) throws IgniteInternalCheckedException {
         return new VersionChainFreeList(
                 FREE_LIST_GROUP_ID,
                 FREE_LIST_PARTITION_ID,
                 pageMemory,
-                reuseList,
                 PageLockListenerNoOp.INSTANCE,
-                metaPageId,
+                pageMemory.allocatePage(FREE_LIST_GROUP_ID, FREE_LIST_PARTITION_ID, FLAG_AUX),
                 true,
-                null,
                 PageEvictionTrackerNoOp.INSTANCE,
                 IoStatisticsHolderNoOp.INSTANCE
         );
     }
 
     private static RowVersionFreeList createRowVersionFreeList(
             PageMemory pageMemory,
-            ReuseList reuseList
+            VersionChainFreeList reuseList
     ) throws IgniteInternalCheckedException {
-        long metaPageId = pageMemory.allocatePage(
-                FREE_LIST_GROUP_ID,
-                FREE_LIST_PARTITION_ID,
-                FLAG_AUX
-        );
-
         return new RowVersionFreeList(
                 FREE_LIST_GROUP_ID,
                 FREE_LIST_PARTITION_ID,
                 pageMemory,
                 reuseList,
                 PageLockListenerNoOp.INSTANCE,
-                metaPageId,
+                pageMemory.allocatePage(FREE_LIST_GROUP_ID, FREE_LIST_PARTITION_ID, FLAG_AUX),

Review Comment:
   And here. Why?



##########
modules/storage-page-memory/src/main/java/org/apache/ignite/internal/storage/pagememory/mv/PageMemoryMvStorageIoModule.java:
##########
@@ -28,7 +28,7 @@
 import org.apache.ignite.internal.storage.pagememory.mv.io.VersionChainMetaIo;
 
 /**
- * {@link PageIoModule} related to {@link PageMemoryMvPartitionStorage} implementation.
+ * {@link PageIoModule} related to {@link AbstractPageMemoryMvPartitionStorage} implementation.

Review Comment:
   Looks weird when it points to an abstract class. I think we can list 2 MvTableStorage implementations here, would be better.



##########
modules/storage-page-memory/src/main/java/org/apache/ignite/internal/storage/pagememory/PersistentPageMemoryTableStorage.java:
##########
@@ -164,8 +170,113 @@ public void destroy() throws StorageException {
 
     /** {@inheritDoc} */
     @Override
-    public PageMemoryMvPartitionStorage createMvPartitionStorage(int partitionId) {
-        throw new UnsupportedOperationException("Not supported yet");
+    public PersistentPageMemoryMvPartitionStorage createMvPartitionStorage(int partitionId) {
+        TableView tableView = tableCfg.value();
+
+        FilePageStore filePageStore = ensurePartitionFilePageStore(tableView, partitionId);
+
+        CheckpointManager checkpointManager = dataRegion.checkpointManager();
+
+        CheckpointTimeoutLock checkpointTimeoutLock = checkpointManager.checkpointTimeoutLock();
+
+        checkpointTimeoutLock.checkpointReadLock();
+
+        try {
+            PersistentPageMemory persistentPageMemory = dataRegion.pageMemory();
+
+            int grpId = tableView.tableId();
+
+            CheckpointProgress lastCheckpointProgress = checkpointManager.lastCheckpointProgress();
+
+            UUID checkpointId = lastCheckpointProgress == null ? null : lastCheckpointProgress.id();
+
+            PartitionMeta meta = dataRegion.partitionMetaManager().readOrCreateMeta(
+                    checkpointId,
+                    new GroupPartitionId(grpId, partitionId),
+                    filePageStore
+            );
+
+            dataRegion.partitionMetaManager().addMeta(new GroupPartitionId(grpId, partitionId), meta);
+
+            filePageStore.pages(meta.pageCount());
+
+            filePageStore.setPageAllocationListener(pageIdx -> {
+                assert checkpointTimeoutLock.checkpointLockIsHeldByThread();
+
+                CheckpointProgress last = checkpointManager.lastCheckpointProgress();
+
+                meta.incrementPageCount(last == null ? null : last.id());
+            });
+
+            boolean initNewVersionChainTree = false;
+
+            if (meta.versionChainTreeRootPageId() == 0) {
+                meta.versionChainTreeRootPageId(checkpointId, persistentPageMemory.allocatePage(grpId, partitionId, FLAG_AUX));
+
+                initNewVersionChainTree = true;
+            }
+
+            boolean initVersionChainFreeList = false;
+
+            if (meta.versionChainFreeListRootPageId() == 0) {
+                meta.versionChainFreeListRootPageId(checkpointId, persistentPageMemory.allocatePage(grpId, partitionId, FLAG_AUX));
+
+                initVersionChainFreeList = true;
+            }
+
+            boolean initRowVersionFreeList = false;
+
+            if (meta.rowVersionFreeListRootPageId() == 0) {
+                meta.rowVersionFreeListRootPageId(checkpointId, persistentPageMemory.allocatePage(grpId, partitionId, FLAG_AUX));
+
+                initRowVersionFreeList = true;
+            }
+
+            VersionChainFreeList versionChainFreeList = createVersionChainFreeList(
+                    tableView,
+                    partitionId,
+                    meta.versionChainFreeListRootPageId(),
+                    initVersionChainFreeList
+            );
+
+            autoCloseables.add(versionChainFreeList::close);
+
+            RowVersionFreeList rowVersionFreeList = createRowVersionFreeList(
+                    tableView,
+                    partitionId,
+                    versionChainFreeList,
+                    meta.rowVersionFreeListRootPageId(),
+                    initRowVersionFreeList
+            );
+
+            autoCloseables.add(rowVersionFreeList::close);
+
+            VersionChainTree versionChainTree = createVersionChainTree(
+                    tableView,
+                    partitionId,
+                    versionChainFreeList,
+                    meta.versionChainTreeRootPageId(),
+                    initNewVersionChainTree
+            );
+
+            return new PersistentPageMemoryMvPartitionStorage(
+                    partitionId,
+                    tableView,
+                    dataRegion.pageMemory(),
+                    versionChainFreeList,
+                    rowVersionFreeList,
+                    versionChainTree,
+                    checkpointTimeoutLock
+            );
+        } catch (IgniteInternalCheckedException e) {
+            throw new StorageException(
+                    String.format("Error getting or creating partition metadata [tableName=%s, partitionId=%s]", tableView.name(),

Review Comment:
   Not the best description. Just remove the word "metadata" and it'll get better



##########
modules/storage-page-memory/src/main/java/org/apache/ignite/internal/storage/pagememory/VolatilePageMemoryDataRegion.java:
##########
@@ -104,60 +103,43 @@ public void start() {
     }
 
     private TableFreeList createTableFreeList(PageMemory pageMemory) throws IgniteInternalCheckedException {
-        long metaPageId = pageMemory.allocatePage(FREE_LIST_GROUP_ID, FREE_LIST_PARTITION_ID, FLAG_AUX);
-
         return new TableFreeList(
                 FREE_LIST_GROUP_ID,
                 FREE_LIST_PARTITION_ID,
                 pageMemory,
                 PageLockListenerNoOp.INSTANCE,
-                metaPageId,
+                pageMemory.allocatePage(FREE_LIST_GROUP_ID, FREE_LIST_PARTITION_ID, FLAG_AUX),
                 true,
-                null,
                 PageEvictionTrackerNoOp.INSTANCE,
                 IoStatisticsHolderNoOp.INSTANCE
         );
     }
 
-    private static VersionChainFreeList createVersionChainFreeList(
-            PageMemory pageMemory,
-            ReuseList reuseList
-    ) throws IgniteInternalCheckedException {
-        long metaPageId = pageMemory.allocatePage(FREE_LIST_GROUP_ID, FREE_LIST_PARTITION_ID, FLAG_AUX);
-
+    private static VersionChainFreeList createVersionChainFreeList(PageMemory pageMemory) throws IgniteInternalCheckedException {
         return new VersionChainFreeList(
                 FREE_LIST_GROUP_ID,
                 FREE_LIST_PARTITION_ID,
                 pageMemory,
-                reuseList,
                 PageLockListenerNoOp.INSTANCE,
-                metaPageId,
+                pageMemory.allocatePage(FREE_LIST_GROUP_ID, FREE_LIST_PARTITION_ID, FLAG_AUX),

Review Comment:
   Same here



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: notifications-unsubscribe@ignite.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org