This is an automated email from the ASF dual-hosted git repository.
shishkovilja pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/ignite.git
The following commit(s) were added to refs/heads/master by this push:
new 12d25c4c110 IGNITE-26569 Refactor GridNearSingleGetRequest,
CacheEntryInfoCollection, GridDistributedTxFinishResponse (#12392)
12d25c4c110 is described below
commit 12d25c4c1108b6d1e162fd8c4a739de33d3543e4
Author: Vladimir Steshin <[email protected]>
AuthorDate: Fri Oct 10 10:51:20 2025 +0300
IGNITE-26569 Refactor GridNearSingleGetRequest, CacheEntryInfoCollection,
GridDistributedTxFinishResponse (#12392)
---
.../communication/GridIoMessageFactory.java | 9 +-
.../processors/cache/CacheEntryInfoCollection.java | 66 +------
.../GridDistributedTxFinishResponse.java | 36 +++-
.../preloader/GridDhtPartitionSupplyMessage.java | 11 +-
.../distributed/near/GridNearSingleGetRequest.java | 216 +++++++--------------
5 files changed, 117 insertions(+), 221 deletions(-)
diff --git
a/modules/core/src/main/java/org/apache/ignite/internal/managers/communication/GridIoMessageFactory.java
b/modules/core/src/main/java/org/apache/ignite/internal/managers/communication/GridIoMessageFactory.java
index d2fd67ae44c..5ed2af4b441 100644
---
a/modules/core/src/main/java/org/apache/ignite/internal/managers/communication/GridIoMessageFactory.java
+++
b/modules/core/src/main/java/org/apache/ignite/internal/managers/communication/GridIoMessageFactory.java
@@ -27,6 +27,7 @@ import org.apache.ignite.internal.GridTaskSessionRequest;
import org.apache.ignite.internal.IgniteDiagnosticMessage;
import
org.apache.ignite.internal.codegen.AtomicApplicationAttributesAwareRequestSerializer;
import
org.apache.ignite.internal.codegen.CacheContinuousQueryBatchAckSerializer;
+import org.apache.ignite.internal.codegen.CacheEntryInfoCollectionSerializer;
import org.apache.ignite.internal.codegen.CacheEvictionEntrySerializer;
import org.apache.ignite.internal.codegen.CacheGroupAffinityMessageSerializer;
import
org.apache.ignite.internal.codegen.CachePartitionPartialCountersMapSerializer;
@@ -54,6 +55,7 @@ import
org.apache.ignite.internal.codegen.GridDhtTxOnePhaseCommitAckRequestSeria
import org.apache.ignite.internal.codegen.GridDhtTxPrepareRequestSerializer;
import org.apache.ignite.internal.codegen.GridDhtUnlockRequestSerializer;
import
org.apache.ignite.internal.codegen.GridDistributedLockResponseSerializer;
+import
org.apache.ignite.internal.codegen.GridDistributedTxFinishResponseSerializer;
import
org.apache.ignite.internal.codegen.GridDistributedTxPrepareRequestSerializer;
import org.apache.ignite.internal.codegen.GridJobCancelRequestSerializer;
import org.apache.ignite.internal.codegen.GridJobSiblingsRequestSerializer;
@@ -61,6 +63,7 @@ import
org.apache.ignite.internal.codegen.GridNearAtomicCheckUpdateRequestSerial
import
org.apache.ignite.internal.codegen.GridNearAtomicUpdateResponseSerializer;
import org.apache.ignite.internal.codegen.GridNearGetRequestSerializer;
import org.apache.ignite.internal.codegen.GridNearLockResponseSerializer;
+import org.apache.ignite.internal.codegen.GridNearSingleGetRequestSerializer;
import org.apache.ignite.internal.codegen.GridNearTxPrepareRequestSerializer;
import org.apache.ignite.internal.codegen.GridNearUnlockRequestSerializer;
import org.apache.ignite.internal.codegen.GridQueryCancelRequestSerializer;
@@ -277,7 +280,7 @@ public class GridIoMessageFactory implements
MessageFactoryProvider {
factory.register((short)21, GridDistributedLockRequest::new);
factory.register((short)22, GridDistributedLockResponse::new, new
GridDistributedLockResponseSerializer());
factory.register((short)23, GridDistributedTxFinishRequest::new);
- factory.register((short)24, GridDistributedTxFinishResponse::new);
+ factory.register((short)24, GridDistributedTxFinishResponse::new, new
GridDistributedTxFinishResponseSerializer());
factory.register((short)25, GridDistributedTxPrepareRequest::new, new
GridDistributedTxPrepareRequestSerializer());
factory.register((short)26, GridDistributedTxPrepareResponse::new);
// Type 27 is former GridDistributedUnlockRequest
@@ -327,7 +330,7 @@ public class GridIoMessageFactory implements
MessageFactoryProvider {
factory.register((short)87, GridDhtPartitionExchangeId::new, new
GridDhtPartitionExchangeIdSerializer());
factory.register((short)88, GridCacheReturn::new);
factory.register((short)91, GridCacheEntryInfo::new, new
GridCacheEntryInfoSerializer());
- factory.register((short)92, CacheEntryInfoCollection::new);
+ factory.register((short)92, CacheEntryInfoCollection::new, new
CacheEntryInfoCollectionSerializer());
factory.register((short)93, CacheInvokeDirectResult::new);
factory.register((short)94, IgniteTxKey::new, new
IgniteTxKeySerializer());
factory.register((short)95, DataStreamerEntry::new);
@@ -348,7 +351,7 @@ public class GridIoMessageFactory implements
MessageFactoryProvider {
// 113 - BinaryObjectImpl
factory.register((short)114, GridDhtPartitionSupplyMessage::new, new
GridDhtPartitionSupplyMessageSerializer());
factory.register((short)115, UUIDCollectionMessage::new);
- factory.register((short)116, GridNearSingleGetRequest::new);
+ factory.register((short)116, GridNearSingleGetRequest::new, new
GridNearSingleGetRequestSerializer());
factory.register((short)117, GridNearSingleGetResponse::new);
factory.register((short)118, CacheContinuousQueryBatchAck::new, new
CacheContinuousQueryBatchAckSerializer());
diff --git
a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/CacheEntryInfoCollection.java
b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/CacheEntryInfoCollection.java
index 2770bf27d2c..2997b9c57e3 100644
---
a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/CacheEntryInfoCollection.java
+++
b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/CacheEntryInfoCollection.java
@@ -17,33 +17,22 @@
package org.apache.ignite.internal.processors.cache;
-import java.nio.ByteBuffer;
import java.util.ArrayList;
import java.util.List;
-import org.apache.ignite.internal.GridDirectCollection;
+import org.apache.ignite.internal.Order;
import org.apache.ignite.plugin.extensions.communication.Message;
-import
org.apache.ignite.plugin.extensions.communication.MessageCollectionItemType;
-import org.apache.ignite.plugin.extensions.communication.MessageReader;
-import org.apache.ignite.plugin.extensions.communication.MessageWriter;
-/**
- *
- */
+/** TODO : IGNITE-26568. Revise, remove. */
public class CacheEntryInfoCollection implements Message {
/** */
- @GridDirectCollection(GridCacheEntryInfo.class)
- private List<GridCacheEntryInfo> infos;
-
- /** */
- public CacheEntryInfoCollection() {
- // No-op
- }
+ @Order(0)
+ private List<GridCacheEntryInfo> infos = new ArrayList<>();
/**
- *
+ * Sets entries.
*/
- public void init() {
- infos = new ArrayList<>();
+ public void infos(List<GridCacheEntryInfo> infos) {
+ this.infos = infos;
}
/**
@@ -60,47 +49,6 @@ public class CacheEntryInfoCollection implements Message {
infos.add(info);
}
- /** {@inheritDoc} */
- @Override public boolean writeTo(ByteBuffer buf, MessageWriter writer) {
- writer.setBuffer(buf);
-
- if (!writer.isHeaderWritten()) {
- if (!writer.writeHeader(directType()))
- return false;
-
- writer.onHeaderWritten();
- }
-
- switch (writer.state()) {
- case 0:
- if (!writer.writeCollection(infos,
MessageCollectionItemType.MSG))
- return false;
-
- writer.incrementState();
-
- }
-
- return true;
- }
-
- /** {@inheritDoc} */
- @Override public boolean readFrom(ByteBuffer buf, MessageReader reader) {
- reader.setBuffer(buf);
-
- switch (reader.state()) {
- case 0:
- infos = reader.readCollection(MessageCollectionItemType.MSG);
-
- if (!reader.isLastRead())
- return false;
-
- reader.incrementState();
-
- }
-
- return true;
- }
-
/** {@inheritDoc} */
@Override public short directType() {
return 92;
diff --git
a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/GridDistributedTxFinishResponse.java
b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/GridDistributedTxFinishResponse.java
index 825745808c8..85059a928bb 100644
---
a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/GridDistributedTxFinishResponse.java
+++
b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/GridDistributedTxFinishResponse.java
@@ -19,6 +19,7 @@ package
org.apache.ignite.internal.processors.cache.distributed;
import java.nio.ByteBuffer;
import org.apache.ignite.IgniteLogger;
+import org.apache.ignite.internal.Order;
import org.apache.ignite.internal.managers.communication.GridIoMessageFactory;
import org.apache.ignite.internal.processors.cache.GridCacheMessage;
import org.apache.ignite.internal.processors.cache.GridCacheSharedContext;
@@ -34,16 +35,20 @@ import
org.apache.ignite.plugin.extensions.communication.MessageWriter;
*/
public class GridDistributedTxFinishResponse extends GridCacheMessage {
/** */
+ @Order(value = 3, method = "xid")
private GridCacheVersion txId;
/** Future ID. */
+ @Order(value = 4, method = "futureId")
private IgniteUuid futId;
- /** */
+ /** TODO: revise this field in IGNITE-26589. Looks like a boolean and is
required only in GridDhtTxFinishResponse. */
@GridToStringExclude
+ @Order(5)
private byte flags;
- /** */
+ /** Partition ID this message is targeted to. */
+ @Order(value = 6, method = "partition")
private int part;
/**
@@ -67,11 +72,26 @@ public class GridDistributedTxFinishResponse extends
GridCacheMessage {
this.futId = futId;
}
+ /** Partition ID this message is targeted to. */
+ public void partition(int part) {
+ this.part = part;
+ }
+
/** {@inheritDoc} */
@Override public final int partition() {
return part;
}
+ /** */
+ public void flags(byte flags) {
+ this.flags = flags;
+ }
+
+ /** */
+ public byte flags() {
+ return flags;
+ }
+
/**
* Sets flag mask.
*
@@ -100,6 +120,16 @@ public class GridDistributedTxFinishResponse extends
GridCacheMessage {
return txId;
}
+ /** */
+ public void xid(GridCacheVersion txId) {
+ this.txId = txId;
+ }
+
+ /** */
+ public void futureId(IgniteUuid futId) {
+ this.futId = futId;
+ }
+
/**
* @return Future ID.
*/
@@ -117,6 +147,7 @@ public class GridDistributedTxFinishResponse extends
GridCacheMessage {
return ctx.txFinishMessageLogger();
}
+ // TODO: remove after IGNITE-26589
/** {@inheritDoc} */
@Override public boolean writeTo(ByteBuffer buf, MessageWriter writer) {
writer.setBuffer(buf);
@@ -161,6 +192,7 @@ public class GridDistributedTxFinishResponse extends
GridCacheMessage {
return true;
}
+ // TODO: remove after IGNITE-26589
/** {@inheritDoc} */
@Override public boolean readFrom(ByteBuffer buf, MessageReader reader) {
reader.setBuffer(buf);
diff --git
a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/preloader/GridDhtPartitionSupplyMessage.java
b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/preloader/GridDhtPartitionSupplyMessage.java
index c01319b763f..e36018e4839 100644
---
a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/preloader/GridDhtPartitionSupplyMessage.java
+++
b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/dht/preloader/GridDhtPartitionSupplyMessage.java
@@ -178,13 +178,8 @@ public class GridDhtPartitionSupplyMessage extends
GridCacheGroupIdMessage imple
msgSize += 12;
// If partition is empty, we need to add it.
- if (!getInfosSafe().containsKey(p)) {
- CacheEntryInfoCollection infoCol = new
CacheEntryInfoCollection();
-
- infoCol.init();
-
- getInfosSafe().put(p, infoCol);
- }
+ if (!getInfosSafe().containsKey(p))
+ getInfosSafe().put(p, new CacheEntryInfoCollection());
}
}
@@ -295,8 +290,6 @@ public class GridDhtPartitionSupplyMessage extends
GridCacheGroupIdMessage imple
msgSize += 4;
getInfosSafe().put(p, infoCol = new CacheEntryInfoCollection());
-
- infoCol.init();
}
infoCol.add(info);
diff --git
a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/near/GridNearSingleGetRequest.java
b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/near/GridNearSingleGetRequest.java
index f3c2bf02281..2d47d8de199 100644
---
a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/near/GridNearSingleGetRequest.java
+++
b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/distributed/near/GridNearSingleGetRequest.java
@@ -17,8 +17,8 @@
package org.apache.ignite.internal.processors.cache.distributed.near;
-import java.nio.ByteBuffer;
import org.apache.ignite.IgniteCheckedException;
+import org.apache.ignite.internal.Order;
import org.apache.ignite.internal.processors.affinity.AffinityTopologyVersion;
import org.apache.ignite.internal.processors.cache.GridCacheContext;
import org.apache.ignite.internal.processors.cache.GridCacheDeployable;
@@ -26,8 +26,6 @@ import
org.apache.ignite.internal.processors.cache.GridCacheIdMessage;
import org.apache.ignite.internal.processors.cache.GridCacheSharedContext;
import org.apache.ignite.internal.processors.cache.KeyCacheObject;
import org.apache.ignite.internal.util.typedef.internal.S;
-import org.apache.ignite.plugin.extensions.communication.MessageReader;
-import org.apache.ignite.plugin.extensions.communication.MessageWriter;
import org.jetbrains.annotations.NotNull;
import org.jetbrains.annotations.Nullable;
@@ -54,27 +52,35 @@ public class GridNearSingleGetRequest extends
GridCacheIdMessage implements Grid
public static final int RECOVERY_FLAG_MASK = 0x20;
/** Future ID. */
+ @Order(value = 4, method = "futureId")
private long futId;
/** */
+ @Order(5)
private KeyCacheObject key;
/** Flags. */
+ @Order(6)
private byte flags;
/** Topology version. */
+ @Order(value = 7, method = "topologyVersion")
private AffinityTopologyVersion topVer;
/** Task name hash. */
+ @Order(8)
private int taskNameHash;
/** TTL for read operation. */
+ @Order(9)
private long createTtl;
/** TTL for read operation. */
+ @Order(10)
private long accessTtl;
/** Transaction label. */
+ @Order(value = 11, method = "txLabel")
private @Nullable String txLbl;
/**
@@ -144,6 +150,13 @@ public class GridNearSingleGetRequest extends
GridCacheIdMessage implements Grid
flags |= RECOVERY_FLAG_MASK;
}
+ /**
+ * Sets the key.
+ */
+ public void key(KeyCacheObject key) {
+ this.key = key;
+ }
+
/**
* @return Key.
*/
@@ -151,6 +164,23 @@ public class GridNearSingleGetRequest extends
GridCacheIdMessage implements Grid
return key;
}
+ /** Sets the flags. */
+ public void flags(byte flags) {
+ this.flags = flags;
+ }
+
+ /** @return Flags. */
+ public byte flags() {
+ return flags;
+ }
+
+ /**
+ * Sets future ID.
+ */
+ public void futureId(long futId) {
+ this.futId = futId;
+ }
+
/**
* @return Future ID.
*/
@@ -158,6 +188,13 @@ public class GridNearSingleGetRequest extends
GridCacheIdMessage implements Grid
return futId;
}
+ /**
+ * Sets task name hash.
+ */
+ public void taskNameHash(int taskNameHash) {
+ this.taskNameHash = taskNameHash;
+ }
+
/**
* Gets task name hash.
*
@@ -167,6 +204,13 @@ public class GridNearSingleGetRequest extends
GridCacheIdMessage implements Grid
return taskNameHash;
}
+ /**
+ * Sets topology version.
+ */
+ public void topologyVersion(AffinityTopologyVersion topVer) {
+ this.topVer = topVer;
+ }
+
/**
* @return Topology version.
*/
@@ -174,6 +218,13 @@ public class GridNearSingleGetRequest extends
GridCacheIdMessage implements Grid
return topVer;
}
+ /**
+ * Sets TTL to set after entry is created, -1 to leave unchanged.
+ */
+ public void createTtl(long createTtl) {
+ this.createTtl = createTtl;
+ }
+
/**
* @return New TTL to set after entry is created, -1 to leave unchanged.
*/
@@ -181,6 +232,13 @@ public class GridNearSingleGetRequest extends
GridCacheIdMessage implements Grid
return createTtl;
}
+ /**
+ * Sets new TTL to set after entry is accessed, -1 to leave unchanged.
+ */
+ public void accessTtl(long accessTtl) {
+ this.accessTtl = accessTtl;
+ }
+
/**
* @return New TTL to set after entry is accessed, -1 to leave unchanged.
*/
@@ -195,6 +253,13 @@ public class GridNearSingleGetRequest extends
GridCacheIdMessage implements Grid
return key.partition();
}
+ /**
+ * Sets the transaction label.
+ */
+ public void txLabel(String txLbl) {
+ this.txLbl = txLbl;
+ }
+
/**
* Get transaction label (may be null).
*
@@ -268,151 +333,6 @@ public class GridNearSingleGetRequest extends
GridCacheIdMessage implements Grid
key.finishUnmarshal(cctx.cacheObjectContext(), ldr);
}
- /** {@inheritDoc} */
- @Override public boolean readFrom(ByteBuffer buf, MessageReader reader) {
- reader.setBuffer(buf);
-
- if (!super.readFrom(buf, reader))
- return false;
-
- switch (reader.state()) {
- case 4:
- accessTtl = reader.readLong();
-
- if (!reader.isLastRead())
- return false;
-
- reader.incrementState();
-
- case 5:
- createTtl = reader.readLong();
-
- if (!reader.isLastRead())
- return false;
-
- reader.incrementState();
-
- case 6:
- flags = reader.readByte();
-
- if (!reader.isLastRead())
- return false;
-
- reader.incrementState();
-
- case 7:
- futId = reader.readLong();
-
- if (!reader.isLastRead())
- return false;
-
- reader.incrementState();
-
- case 8:
- key = reader.readKeyCacheObject();
-
- if (!reader.isLastRead())
- return false;
-
- reader.incrementState();
-
- case 9:
- taskNameHash = reader.readInt();
-
- if (!reader.isLastRead())
- return false;
-
- reader.incrementState();
-
- case 10:
- topVer = reader.readAffinityTopologyVersion();
-
- if (!reader.isLastRead())
- return false;
-
- reader.incrementState();
-
- case 11:
- txLbl = reader.readString();
-
- if (!reader.isLastRead())
- return false;
-
- reader.incrementState();
-
- }
-
- return true;
- }
-
- /** {@inheritDoc} */
- @Override public boolean writeTo(ByteBuffer buf, MessageWriter writer) {
- writer.setBuffer(buf);
-
- if (!super.writeTo(buf, writer))
- return false;
-
- if (!writer.isHeaderWritten()) {
- if (!writer.writeHeader(directType()))
- return false;
-
- writer.onHeaderWritten();
- }
-
- switch (writer.state()) {
- case 4:
- if (!writer.writeLong(accessTtl))
- return false;
-
- writer.incrementState();
-
- case 5:
- if (!writer.writeLong(createTtl))
- return false;
-
- writer.incrementState();
-
- case 6:
- if (!writer.writeByte(flags))
- return false;
-
- writer.incrementState();
-
- case 7:
- if (!writer.writeLong(futId))
- return false;
-
- writer.incrementState();
-
- case 8:
- if (!writer.writeKeyCacheObject(key))
- return false;
-
- writer.incrementState();
-
- case 9:
- if (!writer.writeInt(taskNameHash))
- return false;
-
- writer.incrementState();
-
- case 10:
- if (!writer.writeAffinityTopologyVersion(topVer))
- return false;
-
- writer.incrementState();
-
- case 11:
- if (!writer.writeString(txLbl))
- return false;
-
- writer.incrementState();
-
- }
-
- return true;
- }
-
/** {@inheritDoc} */
@Override public boolean addDeploymentInfo() {
return addDepInfo;