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

vpyatkov 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 abf6e2b30ef IGNITE-26397 OutNetworkObject constructor with 
shouldBeSavedForRecovery is redundant (#6561)
abf6e2b30ef is described below

commit abf6e2b30ef3fd375e428455ecda773b4f1cca4b
Author: Vladislav Pyatkov <[email protected]>
AuthorDate: Mon Sep 8 12:56:50 2025 +0300

    IGNITE-26397 OutNetworkObject constructor with shouldBeSavedForRecovery is 
redundant (#6561)
---
 .../internal/network/netty/ItConnectionManagerTest.java     | 10 +++++-----
 .../apache/ignite/internal/network/OutNetworkObject.java    | 13 +------------
 .../internal/network/netty/InboundRecoveryHandler.java      |  2 +-
 .../internal/network/recovery/HandshakeManagerUtils.java    |  2 +-
 .../network/recovery/RecoveryAcceptorHandshakeManager.java  |  4 ++--
 .../network/recovery/RecoveryInitiatorHandshakeManager.java |  4 ++--
 .../ignite/internal/network/OutNetworkObjectTest.java       |  2 +-
 .../internal/network/recovery/RecoveryDescriptorTest.java   |  6 +++---
 .../disaster/system/SystemDisasterRecoveryManagerImpl.java  |  1 +
 9 files changed, 17 insertions(+), 27 deletions(-)

diff --git 
a/modules/network/src/integrationTest/java/org/apache/ignite/internal/network/netty/ItConnectionManagerTest.java
 
b/modules/network/src/integrationTest/java/org/apache/ignite/internal/network/netty/ItConnectionManagerTest.java
index 33d00b0ec8d..aac41e24895 100644
--- 
a/modules/network/src/integrationTest/java/org/apache/ignite/internal/network/netty/ItConnectionManagerTest.java
+++ 
b/modules/network/src/integrationTest/java/org/apache/ignite/internal/network/netty/ItConnectionManagerTest.java
@@ -407,7 +407,7 @@ public class ItConnectionManagerTest extends 
BaseIgniteAbstractTest {
 
             OutgoingAcknowledgementSilencer ackSilencer = 
dropAcksFrom(manager2);
 
-            CompletableFuture<Void> sendFuture = sender.send(new 
OutNetworkObject(emptyTestMessage, emptyList(), true));
+            CompletableFuture<Void> sendFuture = sender.send(new 
OutNetworkObject(emptyTestMessage, emptyList()));
 
             assertThat(sendFuture, willCompleteSuccessfully());
 
@@ -462,9 +462,9 @@ public class ItConnectionManagerTest extends 
BaseIgniteAbstractTest {
 
             List<Integer> ordinals = new CopyOnWriteArrayList<>();
 
-            CompletableFuture<Void> future1 = sender.send(new 
OutNetworkObject(emptyTestMessage, emptyList(), true))
+            CompletableFuture<Void> future1 = sender.send(new 
OutNetworkObject(emptyTestMessage, emptyList()))
                     .whenComplete((res, ex) -> ordinals.add(1));
-            CompletableFuture<Void> future2 = sender.send(new 
OutNetworkObject(emptyTestMessage, emptyList(), true))
+            CompletableFuture<Void> future2 = sender.send(new 
OutNetworkObject(emptyTestMessage, emptyList()))
                     .whenComplete((res, ex) -> ordinals.add(2));
 
             assertThat(CompletableFuture.allOf(future1, future2), 
willCompleteSuccessfully());
@@ -484,7 +484,7 @@ public class ItConnectionManagerTest extends 
BaseIgniteAbstractTest {
             dropAcksFrom(manager2);
 
             HandshakeFinishMessage messageNotNeedingAck = new 
NetworkMessagesFactory().handshakeFinishMessage().build();
-            CompletableFuture<Void> sendFuture = sender.send(new 
OutNetworkObject(messageNotNeedingAck, emptyList(), true));
+            CompletableFuture<Void> sendFuture = sender.send(new 
OutNetworkObject(messageNotNeedingAck, emptyList()));
             assertThat(sendFuture, willCompleteSuccessfully());
         }
     }
@@ -503,7 +503,7 @@ public class ItConnectionManagerTest extends 
BaseIgniteAbstractTest {
     }
 
     private void provokeAckFor(NettySender sender1) {
-        sender1.send(new OutNetworkObject(emptyTestMessage, emptyList(), 
true));
+        sender1.send(new OutNetworkObject(emptyTestMessage, emptyList()));
     }
 
     /**
diff --git 
a/modules/network/src/main/java/org/apache/ignite/internal/network/OutNetworkObject.java
 
b/modules/network/src/main/java/org/apache/ignite/internal/network/OutNetworkObject.java
index 76373cbf808..c525bb4e231 100644
--- 
a/modules/network/src/main/java/org/apache/ignite/internal/network/OutNetworkObject.java
+++ 
b/modules/network/src/main/java/org/apache/ignite/internal/network/OutNetworkObject.java
@@ -47,20 +47,9 @@ public class OutNetworkObject {
      * @param descriptors Class descriptors.
      */
     public OutNetworkObject(NetworkMessage networkMessage, 
List<ClassDescriptorMessage> descriptors) {
-        this(networkMessage, descriptors, true);
-    }
-
-    /**
-     * Constructor.
-     *
-     * @param networkMessage Network message.
-     * @param descriptors Class descriptors.
-     * @param shouldBeSavedForRecovery See {@link #shouldBeSavedForRecovery}.
-     */
-    public OutNetworkObject(NetworkMessage networkMessage, 
List<ClassDescriptorMessage> descriptors, boolean shouldBeSavedForRecovery) {
         this.networkMessage = networkMessage;
         this.descriptors = descriptors;
-        this.shouldBeSavedForRecovery = shouldBeSavedForRecovery;
+        this.shouldBeSavedForRecovery = networkMessage.needAck();
     }
 
     /**
diff --git 
a/modules/network/src/main/java/org/apache/ignite/internal/network/netty/InboundRecoveryHandler.java
 
b/modules/network/src/main/java/org/apache/ignite/internal/network/netty/InboundRecoveryHandler.java
index 172aa015815..bccaee74db5 100644
--- 
a/modules/network/src/main/java/org/apache/ignite/internal/network/netty/InboundRecoveryHandler.java
+++ 
b/modules/network/src/main/java/org/apache/ignite/internal/network/netty/InboundRecoveryHandler.java
@@ -122,7 +122,7 @@ public class InboundRecoveryHandler extends 
ChannelInboundHandlerAdapter {
         AcknowledgementMessage ackMsg = factory.acknowledgementMessage()
                 .receivedMessages(receiveCnt).build();
 
-        ctx.channel().writeAndFlush(new OutNetworkObject(ackMsg, 
Collections.emptyList(), false));
+        ctx.channel().writeAndFlush(new OutNetworkObject(ackMsg, 
Collections.emptyList()));
 
         lastSentReceivedCount = ackMsg.receivedMessages();
     }
diff --git 
a/modules/network/src/main/java/org/apache/ignite/internal/network/recovery/HandshakeManagerUtils.java
 
b/modules/network/src/main/java/org/apache/ignite/internal/network/recovery/HandshakeManagerUtils.java
index 1998d57d37f..a48226f5d18 100644
--- 
a/modules/network/src/main/java/org/apache/ignite/internal/network/recovery/HandshakeManagerUtils.java
+++ 
b/modules/network/src/main/java/org/apache/ignite/internal/network/recovery/HandshakeManagerUtils.java
@@ -68,7 +68,7 @@ class HandshakeManagerUtils {
                 .message(messageText)
                 .build();
 
-        ChannelFuture sendFuture = channel.writeAndFlush(new 
OutNetworkObject(rejectionMessage, emptyList(), false));
+        ChannelFuture sendFuture = channel.writeAndFlush(new 
OutNetworkObject(rejectionMessage, emptyList()));
 
         NettyUtils.toCompletableFuture(sendFuture).whenComplete((unused, ex) 
-> {
             // Ignoring ex as it's more important to tell the other side about 
the rejection reason.
diff --git 
a/modules/network/src/main/java/org/apache/ignite/internal/network/recovery/RecoveryAcceptorHandshakeManager.java
 
b/modules/network/src/main/java/org/apache/ignite/internal/network/recovery/RecoveryAcceptorHandshakeManager.java
index c07b8a49c32..b20f4be8247 100644
--- 
a/modules/network/src/main/java/org/apache/ignite/internal/network/recovery/RecoveryAcceptorHandshakeManager.java
+++ 
b/modules/network/src/main/java/org/apache/ignite/internal/network/recovery/RecoveryAcceptorHandshakeManager.java
@@ -179,7 +179,7 @@ public class RecoveryAcceptorHandshakeManager implements 
HandshakeManager {
                 
.productVersion(productVersionSource.productVersion().toString())
                 .build();
 
-        ChannelFuture sendFuture = channel.writeAndFlush(new 
OutNetworkObject(handshakeStartMessage, emptyList(), false));
+        ChannelFuture sendFuture = channel.writeAndFlush(new 
OutNetworkObject(handshakeStartMessage, emptyList()));
 
         NettyUtils.toCompletableFuture(sendFuture).whenComplete((unused, 
throwable) -> {
             if (throwable != null) {
@@ -390,7 +390,7 @@ public class RecoveryAcceptorHandshakeManager implements 
HandshakeManager {
                 .build();
 
         CompletableFuture<Void> sendFuture = NettyUtils.toCompletableFuture(
-                channel.write(new OutNetworkObject(response, emptyList(), 
false))
+                channel.write(new OutNetworkObject(response, emptyList()))
         );
 
         descriptor.acknowledge(receivedCount);
diff --git 
a/modules/network/src/main/java/org/apache/ignite/internal/network/recovery/RecoveryInitiatorHandshakeManager.java
 
b/modules/network/src/main/java/org/apache/ignite/internal/network/recovery/RecoveryInitiatorHandshakeManager.java
index dcf85d73b55..1f07fdb4f41 100644
--- 
a/modules/network/src/main/java/org/apache/ignite/internal/network/recovery/RecoveryInitiatorHandshakeManager.java
+++ 
b/modules/network/src/main/java/org/apache/ignite/internal/network/recovery/RecoveryInitiatorHandshakeManager.java
@@ -192,7 +192,7 @@ public class RecoveryInitiatorHandshakeManager implements 
HandshakeManager {
     private void sendProbeToAcceptor() {
         ProbeMessage probe = MESSAGE_FACTORY.probeMessage().build();
 
-        toCompletableFuture(channel.writeAndFlush(new OutNetworkObject(probe, 
List.of(), false))).whenComplete((res, ex) -> {
+        toCompletableFuture(channel.writeAndFlush(new OutNetworkObject(probe, 
List.of()))).whenComplete((res, ex) -> {
             if (ex != null) {
                 if (ex instanceof IOException) {
                     // We don't care: the channel will be reopened.
@@ -485,7 +485,7 @@ public class RecoveryInitiatorHandshakeManager implements 
HandshakeManager {
                 .connectionId(connectionId)
                 .build();
 
-        ChannelFuture sendFuture = ctx.channel().writeAndFlush(new 
OutNetworkObject(response, emptyList(), false));
+        ChannelFuture sendFuture = ctx.channel().writeAndFlush(new 
OutNetworkObject(response, emptyList()));
 
         toCompletableFuture(sendFuture).whenComplete((unused, throwable) -> {
             if (throwable != null) {
diff --git 
a/modules/network/src/test/java/org/apache/ignite/internal/network/OutNetworkObjectTest.java
 
b/modules/network/src/test/java/org/apache/ignite/internal/network/OutNetworkObjectTest.java
index 5932a4367c5..30e2a9e4d1c 100644
--- 
a/modules/network/src/test/java/org/apache/ignite/internal/network/OutNetworkObjectTest.java
+++ 
b/modules/network/src/test/java/org/apache/ignite/internal/network/OutNetworkObjectTest.java
@@ -29,7 +29,7 @@ import 
org.apache.ignite.internal.testframework.BaseIgniteAbstractTest;
 import org.junit.jupiter.api.Test;
 
 class OutNetworkObjectTest extends BaseIgniteAbstractTest {
-    private final OutNetworkObject outObject = new 
OutNetworkObject(mock(NetworkMessage.class), emptyList(), true);
+    private final OutNetworkObject outObject = new 
OutNetworkObject(mock(NetworkMessage.class), emptyList());
 
     @Test
     void acknowledgementFutureIsIncompleteInitially() {
diff --git 
a/modules/network/src/test/java/org/apache/ignite/internal/network/recovery/RecoveryDescriptorTest.java
 
b/modules/network/src/test/java/org/apache/ignite/internal/network/recovery/RecoveryDescriptorTest.java
index 2a51e4b507c..17ce14131f0 100644
--- 
a/modules/network/src/test/java/org/apache/ignite/internal/network/recovery/RecoveryDescriptorTest.java
+++ 
b/modules/network/src/test/java/org/apache/ignite/internal/network/recovery/RecoveryDescriptorTest.java
@@ -146,7 +146,7 @@ class RecoveryDescriptorTest extends BaseIgniteAbstractTest 
{
 
     @Test
     void completesOutObjectFutureOnAcknowledge() {
-        OutNetworkObject outObj = new 
OutNetworkObject(mock(NetworkMessage.class), emptyList(), true);
+        OutNetworkObject outObj = new 
OutNetworkObject(mock(NetworkMessage.class), emptyList());
         descriptor.add(outObj);
 
         descriptor.acknowledge(1);
@@ -156,8 +156,8 @@ class RecoveryDescriptorTest extends BaseIgniteAbstractTest 
{
 
     @Test
     void onlyCompletesFuturesOfAcknowledgedOutObjects() {
-        OutNetworkObject outObj1 = new 
OutNetworkObject(mock(NetworkMessage.class), emptyList(), true);
-        OutNetworkObject outObj2 = new 
OutNetworkObject(mock(NetworkMessage.class), emptyList(), true);
+        OutNetworkObject outObj1 = new 
OutNetworkObject(mock(NetworkMessage.class), emptyList());
+        OutNetworkObject outObj2 = new 
OutNetworkObject(mock(NetworkMessage.class), emptyList());
         descriptor.add(outObj1);
         descriptor.add(outObj2);
 
diff --git 
a/modules/system-disaster-recovery/src/main/java/org/apache/ignite/internal/disaster/system/SystemDisasterRecoveryManagerImpl.java
 
b/modules/system-disaster-recovery/src/main/java/org/apache/ignite/internal/disaster/system/SystemDisasterRecoveryManagerImpl.java
index de2f7f0d0ca..d2b2a59f267 100644
--- 
a/modules/system-disaster-recovery/src/main/java/org/apache/ignite/internal/disaster/system/SystemDisasterRecoveryManagerImpl.java
+++ 
b/modules/system-disaster-recovery/src/main/java/org/apache/ignite/internal/disaster/system/SystemDisasterRecoveryManagerImpl.java
@@ -133,6 +133,7 @@ public class SystemDisasterRecoveryManagerImpl implements 
SystemDisasterRecovery
         restartExecutor.execute(() -> {
             storage.saveResetClusterMessage(message);
 
+            // TODO: IGNITE-26396 A node restart might not send a message to 
the initiator.
             messagingService.respond(sender, successResponseMessage(), 
correlationId)
                     .thenRunAsync(() -> {
                         if (!thisNodeName.equals(sender.name())) {

Reply via email to