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 fc9dff07ae6 IGNITE-27020 Fix CME in outgoing raft snapshots check
(#6939)
fc9dff07ae6 is described below
commit fc9dff07ae68501752a1296ecd939ac7dd7288d6
Author: Ivan Bessonov <[email protected]>
AuthorDate: Tue Nov 11 14:54:30 2025 +0300
IGNITE-27020 Fix CME in outgoing raft snapshots check (#6939)
---
.../raft/snapshot/outgoing/OutgoingSnapshotsManager.java | 5 +++--
.../snapshot/outgoing/OutgoingSnapshotsManagerTest.java | 16 ++++++++++++++--
.../raft/snapshot/SnapshotAwarePartitionDataStorage.java | 12 +++++++++++-
3 files changed, 28 insertions(+), 5 deletions(-)
diff --git
a/modules/partition-replicator/src/main/java/org/apache/ignite/internal/partition/replicator/raft/snapshot/outgoing/OutgoingSnapshotsManager.java
b/modules/partition-replicator/src/main/java/org/apache/ignite/internal/partition/replicator/raft/snapshot/outgoing/OutgoingSnapshotsManager.java
index 9704d144f22..313bd6a5726 100644
---
a/modules/partition-replicator/src/main/java/org/apache/ignite/internal/partition/replicator/raft/snapshot/outgoing/OutgoingSnapshotsManager.java
+++
b/modules/partition-replicator/src/main/java/org/apache/ignite/internal/partition/replicator/raft/snapshot/outgoing/OutgoingSnapshotsManager.java
@@ -33,7 +33,6 @@ import java.util.concurrent.ConcurrentHashMap;
import java.util.concurrent.ExecutorService;
import java.util.concurrent.LinkedBlockingQueue;
import java.util.concurrent.ThreadPoolExecutor;
-import java.util.concurrent.locks.ReadWriteLock;
import java.util.concurrent.locks.ReentrantReadWriteLock;
import org.apache.ignite.internal.failure.FailureContext;
import org.apache.ignite.internal.failure.FailureProcessor;
@@ -251,7 +250,7 @@ public class OutgoingSnapshotsManager implements
PartitionsSnapshots, IgniteComp
private static class PartitionSnapshotsImpl implements PartitionSnapshots {
private final List<OutgoingSnapshot> snapshots = new ArrayList<>();
- private final ReadWriteLock lock = new ReentrantReadWriteLock();
+ private final ReentrantReadWriteLock lock = new
ReentrantReadWriteLock();
private void freezeAndAddUnderLock(OutgoingSnapshot snapshot) {
lock.writeLock().lock();
@@ -290,6 +289,8 @@ public class OutgoingSnapshotsManager implements
PartitionsSnapshots, IgniteComp
@Override
public List<OutgoingSnapshot> ongoingSnapshots() {
+ assert lock.getReadHoldCount() > 0 : "Current thread does not hold
the read lock";
+
return unmodifiableList(snapshots);
}
}
diff --git
a/modules/partition-replicator/src/test/java/org/apache/ignite/internal/partition/replicator/raft/snapshot/outgoing/OutgoingSnapshotsManagerTest.java
b/modules/partition-replicator/src/test/java/org/apache/ignite/internal/partition/replicator/raft/snapshot/outgoing/OutgoingSnapshotsManagerTest.java
index 04d49b5e17a..8bafdceaf0e 100644
---
a/modules/partition-replicator/src/test/java/org/apache/ignite/internal/partition/replicator/raft/snapshot/outgoing/OutgoingSnapshotsManagerTest.java
+++
b/modules/partition-replicator/src/test/java/org/apache/ignite/internal/partition/replicator/raft/snapshot/outgoing/OutgoingSnapshotsManagerTest.java
@@ -71,7 +71,12 @@ class OutgoingSnapshotsManagerTest extends
BaseIgniteAbstractTest {
void emptyOngoingSnapshotsIfNoSnapshotWasRegistered() {
PartitionSnapshots snapshots =
manager.partitionSnapshots(partitionKey);
- assertThat(snapshots.ongoingSnapshots(), is(empty()));
+ snapshots.acquireReadLock();
+ try {
+ assertThat(snapshots.ongoingSnapshots(), is(empty()));
+ } finally {
+ snapshots.releaseReadLock();
+ }
}
@Test
@@ -115,6 +120,13 @@ class OutgoingSnapshotsManagerTest extends
BaseIgniteAbstractTest {
manager.cleanupOutgoingSnapshots(partitionKey);
-
assertThat(manager.partitionSnapshots(partitionKey).ongoingSnapshots(),
is(empty()));
+ PartitionSnapshots snapshots =
manager.partitionSnapshots(partitionKey);
+
+ snapshots.acquireReadLock();
+ try {
+ assertThat(snapshots.ongoingSnapshots(), is(empty()));
+ } finally {
+ snapshots.releaseReadLock();
+ }
}
}
diff --git
a/modules/table/src/main/java/org/apache/ignite/internal/table/distributed/raft/snapshot/SnapshotAwarePartitionDataStorage.java
b/modules/table/src/main/java/org/apache/ignite/internal/table/distributed/raft/snapshot/SnapshotAwarePartitionDataStorage.java
index 57c24eb9d99..6eaeea5b58d 100644
---
a/modules/table/src/main/java/org/apache/ignite/internal/table/distributed/raft/snapshot/SnapshotAwarePartitionDataStorage.java
+++
b/modules/table/src/main/java/org/apache/ignite/internal/table/distributed/raft/snapshot/SnapshotAwarePartitionDataStorage.java
@@ -17,6 +17,8 @@
package org.apache.ignite.internal.table.distributed.raft.snapshot;
+import java.util.ArrayList;
+import java.util.List;
import java.util.UUID;
import java.util.concurrent.CompletableFuture;
import org.apache.ignite.internal.hlc.HybridTimestamp;
@@ -186,9 +188,17 @@ public class SnapshotAwarePartitionDataStorage implements
PartitionDataStorage {
* @param rowId Row id.
*/
private void handleSnapshotInterference(RowId rowId) {
+ List<OutgoingSnapshot> outgoingSnapshots = new ArrayList<>();
+
PartitionSnapshots partitionSnapshots = getPartitionSnapshots();
+ partitionSnapshots.acquireReadLock();
+ try {
+ outgoingSnapshots.addAll(partitionSnapshots.ongoingSnapshots());
+ } finally {
+ partitionSnapshots.releaseReadLock();
+ }
- for (OutgoingSnapshot snapshot :
partitionSnapshots.ongoingSnapshots()) {
+ for (OutgoingSnapshot snapshot : outgoingSnapshots) {
snapshot.acquireMvLock();
try {