This is an automated email from the ASF dual-hosted git repository.

ibessonov pushed a commit to branch main
in repository https://gitbox.apache.org/repos/asf/ignite-3.git


The following commit(s) were added to refs/heads/main by this push:
     new d7027e28497 IGNITE-26996 Use Locker.shouldRelease in MV GC (#6983)
d7027e28497 is described below

commit d7027e28497e1fc48665107b9db9bb9ef8c6d2ac
Author: Viacheslav Blinov <[email protected]>
AuthorDate: Mon Nov 17 15:50:25 2025 +0300

    IGNITE-26996 Use Locker.shouldRelease in MV GC (#6983)
---
 .../storage/impl/TestMvPartitionStorage.java       | 41 ++++++++++++-
 .../table/distributed/gc/GcUpdateHandler.java      | 16 ++++-
 .../gc/AbstractGcUpdateHandlerTest.java            | 68 ++++++++++++++++++++++
 3 files changed, 121 insertions(+), 4 deletions(-)

diff --git 
a/modules/storage-api/src/testFixtures/java/org/apache/ignite/internal/storage/impl/TestMvPartitionStorage.java
 
b/modules/storage-api/src/testFixtures/java/org/apache/ignite/internal/storage/impl/TestMvPartitionStorage.java
index 90b3afc3988..b75bc31b249 100644
--- 
a/modules/storage-api/src/testFixtures/java/org/apache/ignite/internal/storage/impl/TestMvPartitionStorage.java
+++ 
b/modules/storage-api/src/testFixtures/java/org/apache/ignite/internal/storage/impl/TestMvPartitionStorage.java
@@ -34,6 +34,7 @@ import java.util.concurrent.ConcurrentSkipListMap;
 import java.util.concurrent.ConcurrentSkipListSet;
 import java.util.concurrent.atomic.AtomicInteger;
 import java.util.concurrent.atomic.AtomicLongFieldUpdater;
+import java.util.function.BooleanSupplier;
 import org.apache.ignite.internal.hlc.HybridTimestamp;
 import org.apache.ignite.internal.schema.BinaryRow;
 import org.apache.ignite.internal.storage.AbortResult;
@@ -41,6 +42,7 @@ import 
org.apache.ignite.internal.storage.AddWriteCommittedResult;
 import org.apache.ignite.internal.storage.AddWriteResult;
 import org.apache.ignite.internal.storage.CommitResult;
 import org.apache.ignite.internal.storage.MvPartitionStorage;
+import org.apache.ignite.internal.storage.MvPartitionStorage.Locker;
 import org.apache.ignite.internal.storage.PartitionTimestampCursor;
 import org.apache.ignite.internal.storage.ReadResult;
 import org.apache.ignite.internal.storage.RowId;
@@ -97,17 +99,39 @@ public class TestMvPartitionStorage implements 
MvPartitionStorage {
 
     private final LockByRowId lockByRowId;
 
+    private final BooleanSupplier shouldReleaseSupplier;
+
     /** Amount of cursors that opened and still do not close. */
     private final AtomicInteger pendingCursors = new AtomicInteger();
 
     public TestMvPartitionStorage(int partitionId) {
-        this.partitionId = partitionId;
-        this.lockByRowId = new LockByRowId();
+        this(partitionId, new LockByRowId());
     }
 
     public TestMvPartitionStorage(int partitionId, LockByRowId lockByRowId) {
+        this(partitionId, lockByRowId, () -> false);
+    }
+
+    /**
+     * This constructor allows for creating a test partition storage with 
custom lock release behavior,
+     * which is useful for testing scenarios where lock contention needs to be 
simulated (e.g., during
+     * rebalancing or when other operations need to acquire locks held by 
long-running operations like GC).
+     *
+     * @param partitionId Partition ID.
+     * @param lockByRowId Shared lock manager for row-level locking.
+     * @param shouldReleaseSupplier Supplier that determines when locks should 
be released. When this supplier
+     *        returns {@code true}, operations holding locks (like GC vacuum) 
should exit early to allow
+     *        other operations to proceed. See
+     *        {@link Locker#shouldRelease()} for more details.
+     */
+    public TestMvPartitionStorage(
+            int partitionId,
+            LockByRowId lockByRowId,
+            BooleanSupplier shouldReleaseSupplier
+    ) {
         this.partitionId = partitionId;
         this.lockByRowId = lockByRowId;
+        this.shouldReleaseSupplier = shouldReleaseSupplier;
     }
 
     private static class VersionChain implements GcEntry {
@@ -178,7 +202,7 @@ public class TestMvPartitionStorage implements 
MvPartitionStorage {
         if (locker != null) {
             return closure.execute(locker);
         } else {
-            locker = new LocalLocker(lockByRowId);
+            locker = new TestStorageLocker();
 
             THREAD_LOCAL_LOCKER.set(locker);
 
@@ -927,4 +951,15 @@ public class TestMvPartitionStorage implements 
MvPartitionStorage {
 
         return next == null ? null : next.ts;
     }
+
+    private class TestStorageLocker extends LocalLocker {
+        private TestStorageLocker() {
+            super(lockByRowId);
+        }
+
+        @Override
+        public boolean shouldRelease() {
+            return shouldReleaseSupplier.getAsBoolean();
+        }
+    }
 }
diff --git 
a/modules/table/src/main/java/org/apache/ignite/internal/table/distributed/gc/GcUpdateHandler.java
 
b/modules/table/src/main/java/org/apache/ignite/internal/table/distributed/gc/GcUpdateHandler.java
index c0707b94ad0..4fb0358565e 100644
--- 
a/modules/table/src/main/java/org/apache/ignite/internal/table/distributed/gc/GcUpdateHandler.java
+++ 
b/modules/table/src/main/java/org/apache/ignite/internal/table/distributed/gc/GcUpdateHandler.java
@@ -92,6 +92,10 @@ public class GcUpdateHandler {
                         continue;
                     }
 
+                    return true;
+                case SHOULD_RELEASE:
+                    // Storage engine needs resources (e.g., checkpoint needs 
write lock).
+                    // Exit the closure to allow the engine to proceed.
                     return true;
                 default:
                     throw new IllegalStateException(vacuumResult.toString());
@@ -106,6 +110,11 @@ public class GcUpdateHandler {
             int count = countHolder.get();
 
             for (int i = 0; i < count; i++) {
+                // Check if the storage engine needs resources before 
continuing.
+                if (locker.shouldRelease()) {
+                    return VacuumResult.SHOULD_RELEASE;
+                }
+
                 // It is safe for the first iteration to use a lock instead of 
tryLock, since there will be no deadlock for the first RowId
                 // and a deadlock may happen with subsequent ones.
                 VacuumResult vacuumResult = internalVacuum(lowWatermark, 
locker, i > 0);
@@ -123,6 +132,11 @@ public class GcUpdateHandler {
 
     private VacuumResult internalVacuum(HybridTimestamp lowWatermark, Locker 
locker, boolean useTryLock) {
         while (true) {
+            // Check if the storage engine needs resources before continuing.
+            if (locker.shouldRelease()) {
+                return VacuumResult.SHOULD_RELEASE;
+            }
+
             GcEntry gcEntry = storage.peek(lowWatermark);
 
             if (gcEntry == null) {
@@ -156,6 +170,6 @@ public class GcUpdateHandler {
     }
 
     private enum VacuumResult {
-        SUCCESS, NO_GARBAGE_LEFT, FAILED_ACQUIRE_LOCK
+        SUCCESS, NO_GARBAGE_LEFT, FAILED_ACQUIRE_LOCK, SHOULD_RELEASE
     }
 }
diff --git 
a/modules/table/src/test/java/org/apache/ignite/internal/table/distributed/gc/AbstractGcUpdateHandlerTest.java
 
b/modules/table/src/test/java/org/apache/ignite/internal/table/distributed/gc/AbstractGcUpdateHandlerTest.java
index 2a5f5df72a1..c7bb5da96c7 100644
--- 
a/modules/table/src/test/java/org/apache/ignite/internal/table/distributed/gc/AbstractGcUpdateHandlerTest.java
+++ 
b/modules/table/src/test/java/org/apache/ignite/internal/table/distributed/gc/AbstractGcUpdateHandlerTest.java
@@ -25,14 +25,17 @@ import static org.junit.jupiter.api.Assertions.assertTrue;
 import static org.mockito.ArgumentMatchers.any;
 import static org.mockito.ArgumentMatchers.eq;
 import static org.mockito.ArgumentMatchers.isNull;
+import static org.mockito.Mockito.atLeastOnce;
 import static org.mockito.Mockito.clearInvocations;
 import static org.mockito.Mockito.spy;
 import static org.mockito.Mockito.times;
 import static org.mockito.Mockito.verify;
+import static org.mockito.Mockito.when;
 
 import java.util.List;
 import java.util.Map;
 import java.util.UUID;
+import java.util.function.BooleanSupplier;
 import java.util.stream.IntStream;
 import org.apache.ignite.distributed.TestPartitionDataStorage;
 import org.apache.ignite.internal.hlc.HybridTimestamp;
@@ -43,6 +46,9 @@ import org.apache.ignite.internal.storage.MvPartitionStorage;
 import org.apache.ignite.internal.storage.ReadResult;
 import org.apache.ignite.internal.storage.RowId;
 import org.apache.ignite.internal.storage.engine.MvTableStorage;
+import org.apache.ignite.internal.storage.impl.TestMvPartitionStorage;
+import org.apache.ignite.internal.storage.util.LocalLocker;
+import org.apache.ignite.internal.storage.util.LockByRowId;
 import org.apache.ignite.internal.table.distributed.index.IndexUpdateHandler;
 import org.apache.ignite.internal.table.impl.DummyInternalTableImpl;
 import org.apache.ignite.internal.util.Cursor;
@@ -52,6 +58,7 @@ import org.junit.jupiter.api.RepeatedTest;
 import org.junit.jupiter.api.Test;
 import org.junit.jupiter.params.ParameterizedTest;
 import org.junit.jupiter.params.provider.ValueSource;
+import org.mockito.Mockito;
 
 /**
  * Abstract class for testing {@link GcUpdateHandler} using different 
implementations of {@link MvPartitionStorage}.
@@ -250,10 +257,71 @@ abstract class AbstractGcUpdateHandlerTest extends 
BaseMvStoragesTest {
         }
     }
 
+    /**
+     * Tests that {@link GcUpdateHandler#vacuumBatch} exits early when {@code 
shouldRelease()} returns {@code true}.
+     *
+     * <p>This test verifies that the vacuum process can be interrupted 
mid-execution by the {@code shouldRelease()}
+     * mechanism, which is used to allow other operations (like rebalancing) 
to acquire necessary locks.
+     */
+    @Test
+    void testShouldReleaseExitsEarly() {
+        // Given: A partition storage with controllable shouldRelease() 
behavior
+        BooleanSupplier shouldReleaseSupplier = Mockito.mock();
+        TestPartitionDataStorage partitionStorage = 
createTestMvPartitionStorage(shouldReleaseSupplier);
+        IndexUpdateHandler indexUpdateHandler = createIndexUpdateHandler();
+        GcUpdateHandler gcUpdateHandler = 
createGcUpdateHandler(partitionStorage, indexUpdateHandler);
+
+        HybridTimestamp lowWatermark = HybridTimestamp.MAX_VALUE;
+
+        // Given: Multiple rows with garbage to collect (20 rows with 2 
versions each)
+        for (int i = 0; i < 10; i++) {
+            RowId rowId = new RowId(PARTITION_ID);
+            BinaryRow row = binaryRow(new TestKey(i, "key" + i), new 
TestValue(i, "value" + i));
+
+            addWriteCommitted(partitionStorage, rowId, row, clock.now());
+            addWriteCommitted(partitionStorage, rowId, row, clock.now());
+        }
+
+        // When: Vacuum runs with shouldRelease() returning true (simulating 
lock contention)
+        when(shouldReleaseSupplier.getAsBoolean()).thenReturn(true);
+        boolean hasGarbageLeft = gcUpdateHandler.vacuumBatch(lowWatermark, 10, 
true);
+
+        // Then: Vacuum exits early and reports garbage remaining
+        assertTrue(hasGarbageLeft, "Expected garbage to remain after early 
exit");
+        verify(shouldReleaseSupplier).getAsBoolean();
+
+        // When: Vacuum runs again with shouldRelease() returning false
+        when(shouldReleaseSupplier.getAsBoolean()).thenReturn(false);
+        hasGarbageLeft = gcUpdateHandler.vacuumBatch(lowWatermark, 100, true);
+
+        // Then: All remaining garbage is processed
+        assertFalse(hasGarbageLeft, "Expected no garbage left after completing 
vacuum");
+        verify(shouldReleaseSupplier, atLeastOnce()).getAsBoolean();
+    }
+
     private TestPartitionDataStorage createPartitionDataStorage() {
         return new TestPartitionDataStorage(TABLE_ID, PARTITION_ID, 
getOrCreateMvPartition(tableStorage, PARTITION_ID));
     }
 
+    /**
+     * Creates a partition data storage with custom {@code shouldRelease()} 
behavior.
+     *
+     * <p>The returned storage uses a {@link LocalLocker} that checks the 
provided {@code shouldReleaseSupplier}
+     * during lock acquisition, allowing tests to simulate lock contention 
scenarios.
+     *
+     * @param shouldReleaseSupplier Supplier that determines when locks should 
be released.
+     * @return A test partition storage with controllable shouldRelease 
behavior.
+     */
+    private static TestPartitionDataStorage createTestMvPartitionStorage(
+            BooleanSupplier shouldReleaseSupplier
+    ) {
+        LockByRowId lockByRowId = new LockByRowId();
+
+        TestMvPartitionStorage mvStorage = new 
TestMvPartitionStorage(PARTITION_ID, lockByRowId, shouldReleaseSupplier);
+
+        return new TestPartitionDataStorage(TABLE_ID, PARTITION_ID, mvStorage);
+    }
+
     private static IndexUpdateHandler createIndexUpdateHandler() {
         // Don’t use mocking to avoid performance degradation for concurrent 
tests.
         return new 
IndexUpdateHandler(DummyInternalTableImpl.createTableIndexStoragesSupplier(Map.of()))
 {

Reply via email to