You are viewing a plain text version of this content. The canonical link for it is here.
Posted to notifications@geode.apache.org by GitBox <gi...@apache.org> on 2022/04/12 21:27:44 UTC

[GitHub] [geode] pivotal-jbarrett commented on a diff in pull request #7323: GEODE-9997: added ParallelQueueSetPossibleDuplicateMessage

pivotal-jbarrett commented on code in PR #7323:
URL: https://github.com/apache/geode/pull/7323#discussion_r848873170


##########
geode-core/src/main/java/org/apache/geode/internal/cache/AbstractBucketRegionQueue.java:
##########
@@ -223,7 +231,12 @@ protected void loadEventsFromTempQueue() {
       // .getBucketToTempQueueMap().get(getId());
       if (tempQueue != null && !tempQueue.isEmpty()) {
         synchronized (tempQueue) {
+          Map<String, Map<Integer, List<Object>>> regionToDuplicateEventsMap =
+              new ConcurrentHashMap<>();
           try {
+            boolean notifyDuplicate =

Review Comment:
   Extract to a method, say `boolean notifyDuplicateSupported()`, for readability.



##########
geode-core/src/main/java/org/apache/geode/internal/cache/AbstractBucketRegionQueue.java:
##########
@@ -257,11 +273,45 @@ protected void loadEventsFromTempQueue() {
             }
             getInitializationLock().writeLock().unlock();
           }
+          if (regionToDuplicateEventsMap.size() > 0

Review Comment:
   Can this new phase of of this method be extracted into a new method?



##########
geode-core/src/main/java/org/apache/geode/internal/cache/wan/AbstractGatewaySenderEventProcessor.java:
##########
@@ -1322,17 +1325,180 @@ protected void afterExecute() {
 
   protected abstract void enqueueEvent(GatewayQueueEvent<?, ?> event);
 
-  private void pendingEventsInBatchesMarkAsPossibleDuplicate() {
+  private void notifyPossibleDuplicate(int reason, List<?> events) {
+    Map<String, Map<Integer, List<Object>>> regionToDispatchedKeysMap = new ConcurrentHashMap<>();
+    boolean pgwsender = (getSender().isParallel()
+        && !(getDispatcher() instanceof GatewaySenderEventCallbackDispatcher));
+
+    for (Object o : events) {
+      if (o instanceof GatewaySenderEventImpl) {
+        GatewaySenderEventImpl ge = (GatewaySenderEventImpl) o;
+        if (!ge.getPossibleDuplicate()) {
+          if (pgwsender) {
+            addDuplicateEvent(regionToDispatchedKeysMap, ge);
+          }
+          ge.setPossibleDuplicate(true);
+        }
+      }
+    }
+
+    if (!pgwsender) {
+      return;
+    }
+
+    PartitionedRegion queueRegion;
+    if (queue instanceof ConcurrentParallelGatewaySenderQueue) {
+      queueRegion =
+          (PartitionedRegion) ((ConcurrentParallelGatewaySenderQueue) queue).getRegion();
+    } else {
+      queueRegion =
+          (PartitionedRegion) ((ParallelGatewaySenderQueue) queue).getRegion();
+    }
+
+    if (queueRegion == null || queueRegion.getRegionAdvisor() == null
+        || queueRegion.getDataStore() == null) {
+      return;
+    }
+
+    if (reason == STOPPED_GATEWAY_SENDER) {
+      final Set<Integer> buckets = queueRegion.getDataStore().getAllLocalPrimaryBucketIds();
+      if (regionToDispatchedKeysMap.isEmpty()) {
+        if (queueRegion.isSentGatewaySenderStoppedMessage()) {
+          return;
+        }
+        Map<Integer, List<Object>> bucketIdToDispatchedKeys = new ConcurrentHashMap<>();
+        for (Integer bId : buckets) {
+          bucketIdToDispatchedKeys.put(bId, Collections.emptyList());

Review Comment:
   Is it really necessary to populate this map with empty lists? Can `null` be sufficient?



##########
geode-core/src/main/java/org/apache/geode/internal/cache/AbstractBucketRegionQueue.java:
##########
@@ -257,11 +273,45 @@ protected void loadEventsFromTempQueue() {
             }
             getInitializationLock().writeLock().unlock();
           }
+          if (regionToDuplicateEventsMap.size() > 0
+              && getPartitionedRegion().getRegionAdvisor() != null) {
+            Set<InternalDistributedMember> recipients =
+                getPartitionedRegion().getRegionAdvisor().adviseDataStore();
+
+            if (recipients.isEmpty()) {
+              return;
+            }
+
+            InternalDistributedSystem ids = getCache().getInternalDistributedSystem();
+            DistributionManager dm = ids.getDistributionManager();
+            dm.retainMembersWithSameOrNewerVersion(recipients, KnownVersion.GEODE_1_15_0);
+
+            if (!recipients.isEmpty()) {
+              ParallelQueueSetPossibleDuplicateMessage possibleDuplicateMessage =
+                  new ParallelQueueSetPossibleDuplicateMessage(LOAD_FROM_TEMP_QUEUE,
+                      regionToDuplicateEventsMap);
+              possibleDuplicateMessage.setRecipients(recipients);
+              dm.putOutgoing(possibleDuplicateMessage);
+            }
+          }
         }
       }
+    }
+  }
 
-      // }
+  private void addDuplicateEvent(Map<String, Map<Integer, List<Object>>> regionToDispatchedKeysMap,
+      GatewaySenderEventImpl event) {
+    Map<Integer, List<Object>> bucketIdToDispatchedKeys =
+        regionToDispatchedKeysMap.get(getPartitionedRegion().getFullPath());
+    if (bucketIdToDispatchedKeys == null) {

Review Comment:
   Replace with `computeIfAbsent()`.



##########
geode-core/src/main/java/org/apache/geode/internal/cache/wan/AbstractGatewaySender.java:
##########
@@ -1298,6 +1301,26 @@ public boolean removeFromTempQueueEvents(Object tailKey) {
     }
   }
 
+  public boolean markAsDuplicateInTempQueueEvents(Object tailKey) {
+    synchronized (queuedEventsSync) {
+      Iterator<TmpQueueEvent> itr = tmpQueuedEvents.iterator();
+      while (itr.hasNext()) {
+        TmpQueueEvent event = itr.next();
+        if (tailKey.equals(event.getEvent().getTailKey())) {
+          if (logger.isDebugEnabled()) {

Review Comment:
   Check for if debug enabled once outside this loop and use a final variable here to gate the logging.



##########
geode-core/src/main/java/org/apache/geode/internal/cache/wan/AbstractGatewaySender.java:
##########
@@ -1298,6 +1301,26 @@ public boolean removeFromTempQueueEvents(Object tailKey) {
     }
   }
 
+  public boolean markAsDuplicateInTempQueueEvents(Object tailKey) {
+    synchronized (queuedEventsSync) {
+      Iterator<TmpQueueEvent> itr = tmpQueuedEvents.iterator();
+      while (itr.hasNext()) {

Review Comment:
   For each loop?



##########
geode-core/src/main/java/org/apache/geode/internal/cache/wan/parallel/ParallelQueueSetPossibleDuplicateMessage.java:
##########
@@ -0,0 +1,166 @@
+/*
+ * 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.geode.internal.cache.wan.parallel;
+
+import static org.apache.geode.cache.Region.SEPARATOR;
+import static org.apache.geode.internal.cache.LocalRegion.InitializationLevel.BEFORE_INITIAL_IMAGE;
+
+import java.io.DataInput;
+import java.io.DataOutput;
+import java.io.IOException;
+import java.util.List;
+import java.util.Map;
+
+import org.apache.logging.log4j.Logger;
+
+import org.apache.geode.DataSerializer;
+import org.apache.geode.distributed.internal.ClusterDistributionManager;
+import org.apache.geode.distributed.internal.PooledDistributionMessage;
+import org.apache.geode.internal.cache.BucketRegionQueue;
+import org.apache.geode.internal.cache.InternalCache;
+import org.apache.geode.internal.cache.LocalRegion;
+import org.apache.geode.internal.cache.LocalRegion.InitializationLevel;
+import org.apache.geode.internal.cache.PartitionedRegion;
+import org.apache.geode.internal.cache.PartitionedRegionHelper;
+import org.apache.geode.internal.cache.wan.AbstractGatewaySender;
+import org.apache.geode.internal.serialization.DeserializationContext;
+import org.apache.geode.internal.serialization.SerializationContext;
+import org.apache.geode.logging.internal.log4j.api.LogService;
+
+/**
+ * Sets events in the remote secondary queues to possible duplicate
+ *
+ * @since Geode 1.15
+ */
+public class ParallelQueueSetPossibleDuplicateMessage extends PooledDistributionMessage {
+
+  private static final Logger logger = LogService.getLogger();
+
+  private int reason;
+  private Map regionToDispatchedKeysMap;

Review Comment:
   Use type parameters.



##########
geode-core/src/main/java/org/apache/geode/internal/cache/wan/parallel/ParallelQueueSetPossibleDuplicateMessage.java:
##########
@@ -0,0 +1,166 @@
+/*
+ * 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.geode.internal.cache.wan.parallel;
+
+import static org.apache.geode.cache.Region.SEPARATOR;
+import static org.apache.geode.internal.cache.LocalRegion.InitializationLevel.BEFORE_INITIAL_IMAGE;
+
+import java.io.DataInput;
+import java.io.DataOutput;
+import java.io.IOException;
+import java.util.List;
+import java.util.Map;
+
+import org.apache.logging.log4j.Logger;
+
+import org.apache.geode.DataSerializer;
+import org.apache.geode.distributed.internal.ClusterDistributionManager;
+import org.apache.geode.distributed.internal.PooledDistributionMessage;
+import org.apache.geode.internal.cache.BucketRegionQueue;
+import org.apache.geode.internal.cache.InternalCache;
+import org.apache.geode.internal.cache.LocalRegion;
+import org.apache.geode.internal.cache.LocalRegion.InitializationLevel;
+import org.apache.geode.internal.cache.PartitionedRegion;
+import org.apache.geode.internal.cache.PartitionedRegionHelper;
+import org.apache.geode.internal.cache.wan.AbstractGatewaySender;
+import org.apache.geode.internal.serialization.DeserializationContext;
+import org.apache.geode.internal.serialization.SerializationContext;
+import org.apache.geode.logging.internal.log4j.api.LogService;
+
+/**
+ * Sets events in the remote secondary queues to possible duplicate
+ *
+ * @since Geode 1.15
+ */
+public class ParallelQueueSetPossibleDuplicateMessage extends PooledDistributionMessage {
+
+  private static final Logger logger = LogService.getLogger();
+
+  private int reason;
+  private Map regionToDispatchedKeysMap;
+
+  public static final int UNSUCCESSFULLY_DISPATCHED = 0;
+  public static final int RESET_BATCH = 1;
+  public static final int LOAD_FROM_TEMP_QUEUE = 2;
+  public static final int STOPPED_GATEWAY_SENDER = 3;
+
+
+  public ParallelQueueSetPossibleDuplicateMessage() {}
+
+  public ParallelQueueSetPossibleDuplicateMessage(int reason, Map rgnToDispatchedKeysMap) {
+    this.reason = reason;
+    this.regionToDispatchedKeysMap = rgnToDispatchedKeysMap;
+  }
+
+  @Override
+  public int getDSFID() {
+    return PARALLEL_QUEUE_SET_POSSIBLE_DUPLICATE_MESSAGE;
+  }
+
+  @Override
+  public String toString() {
+    String cname = getShortClassName();
+    final StringBuilder sb = new StringBuilder(cname);
+    sb.append("reason=" + reason);
+    sb.append(" regionToDispatchedKeysMap=" + regionToDispatchedKeysMap);
+    sb.append(" sender=").append(getSender());
+    return sb.toString();
+  }
+
+  @Override
+  protected void process(ClusterDistributionManager dm) {
+    final boolean isDebugEnabled = logger.isDebugEnabled();
+    final InternalCache cache = dm.getCache();
+
+    if (cache != null) {
+      final InitializationLevel oldLevel =
+          LocalRegion.setThreadInitLevelRequirement(BEFORE_INITIAL_IMAGE);
+      try {
+        for (Object name : regionToDispatchedKeysMap.keySet()) {
+          final String regionName = (String) name;
+          final PartitionedRegion region = (PartitionedRegion) cache.getRegion(regionName);
+          if (region == null) {
+            continue;
+          } else {
+            AbstractGatewaySender abstractSender = region.getParallelGatewaySender();
+            // Find the map: bucketId to dispatchedKeys
+            // Find the bucket
+            // Destroy the keys
+            Map bucketIdToDispatchedKeys = (Map) this.regionToDispatchedKeysMap.get(regionName);
+            for (Object bId : bucketIdToDispatchedKeys.keySet()) {
+              final String bucketFullPath =
+                  SEPARATOR + PartitionedRegionHelper.PR_ROOT_REGION_NAME + SEPARATOR
+                      + region.getBucketName((Integer) bId);
+              BucketRegionQueue brq =
+                  (BucketRegionQueue) cache.getInternalRegionByPath(bucketFullPath);
+
+              if (brq != null) {
+                if (reason == STOPPED_GATEWAY_SENDER) {
+                  brq.setReceivedGatewaySenderStoppedMessage(true);
+                }
+              }
+
+              if (isDebugEnabled) {
+                logger.debug(
+                    "ParallelQueueSetPossibleDuplicateMessage : The bucket in the cache is bucketRegionName : {} bucket: {}",
+                    bucketFullPath, brq);
+              }
+
+              List dispatchedKeys = (List) bucketIdToDispatchedKeys.get((Integer) bId);

Review Comment:
   Explicit boxing is not required.
   Use type parameters.



##########
geode-core/src/main/java/org/apache/geode/internal/cache/wan/parallel/ParallelQueueSetPossibleDuplicateMessage.java:
##########
@@ -0,0 +1,166 @@
+/*
+ * 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.geode.internal.cache.wan.parallel;
+
+import static org.apache.geode.cache.Region.SEPARATOR;
+import static org.apache.geode.internal.cache.LocalRegion.InitializationLevel.BEFORE_INITIAL_IMAGE;
+
+import java.io.DataInput;
+import java.io.DataOutput;
+import java.io.IOException;
+import java.util.List;
+import java.util.Map;
+
+import org.apache.logging.log4j.Logger;
+
+import org.apache.geode.DataSerializer;
+import org.apache.geode.distributed.internal.ClusterDistributionManager;
+import org.apache.geode.distributed.internal.PooledDistributionMessage;
+import org.apache.geode.internal.cache.BucketRegionQueue;
+import org.apache.geode.internal.cache.InternalCache;
+import org.apache.geode.internal.cache.LocalRegion;
+import org.apache.geode.internal.cache.LocalRegion.InitializationLevel;
+import org.apache.geode.internal.cache.PartitionedRegion;
+import org.apache.geode.internal.cache.PartitionedRegionHelper;
+import org.apache.geode.internal.cache.wan.AbstractGatewaySender;
+import org.apache.geode.internal.serialization.DeserializationContext;
+import org.apache.geode.internal.serialization.SerializationContext;
+import org.apache.geode.logging.internal.log4j.api.LogService;
+
+/**
+ * Sets events in the remote secondary queues to possible duplicate
+ *
+ * @since Geode 1.15
+ */
+public class ParallelQueueSetPossibleDuplicateMessage extends PooledDistributionMessage {
+
+  private static final Logger logger = LogService.getLogger();
+
+  private int reason;
+  private Map regionToDispatchedKeysMap;
+
+  public static final int UNSUCCESSFULLY_DISPATCHED = 0;
+  public static final int RESET_BATCH = 1;
+  public static final int LOAD_FROM_TEMP_QUEUE = 2;
+  public static final int STOPPED_GATEWAY_SENDER = 3;
+
+
+  public ParallelQueueSetPossibleDuplicateMessage() {}
+
+  public ParallelQueueSetPossibleDuplicateMessage(int reason, Map rgnToDispatchedKeysMap) {
+    this.reason = reason;
+    this.regionToDispatchedKeysMap = rgnToDispatchedKeysMap;
+  }
+
+  @Override
+  public int getDSFID() {
+    return PARALLEL_QUEUE_SET_POSSIBLE_DUPLICATE_MESSAGE;
+  }
+
+  @Override
+  public String toString() {
+    String cname = getShortClassName();
+    final StringBuilder sb = new StringBuilder(cname);
+    sb.append("reason=" + reason);
+    sb.append(" regionToDispatchedKeysMap=" + regionToDispatchedKeysMap);
+    sb.append(" sender=").append(getSender());
+    return sb.toString();
+  }
+
+  @Override
+  protected void process(ClusterDistributionManager dm) {
+    final boolean isDebugEnabled = logger.isDebugEnabled();
+    final InternalCache cache = dm.getCache();
+
+    if (cache != null) {

Review Comment:
   Return if `cache` is `null` to reduce nesting, early exit.



##########
geode-core/src/main/java/org/apache/geode/internal/cache/wan/AbstractGatewaySenderEventProcessor.java:
##########
@@ -1322,17 +1325,180 @@ protected void afterExecute() {
 
   protected abstract void enqueueEvent(GatewayQueueEvent<?, ?> event);
 
-  private void pendingEventsInBatchesMarkAsPossibleDuplicate() {
+  private void notifyPossibleDuplicate(int reason, List<?> events) {
+    Map<String, Map<Integer, List<Object>>> regionToDispatchedKeysMap = new ConcurrentHashMap<>();

Review Comment:
   Why `ConcurrentHashMap`? It doesn't seem to escape this stack/thread so where is the concurrency?



##########
geode-core/src/main/java/org/apache/geode/internal/cache/wan/parallel/ParallelQueueSetPossibleDuplicateMessage.java:
##########
@@ -0,0 +1,166 @@
+/*
+ * 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.geode.internal.cache.wan.parallel;
+
+import static org.apache.geode.cache.Region.SEPARATOR;
+import static org.apache.geode.internal.cache.LocalRegion.InitializationLevel.BEFORE_INITIAL_IMAGE;
+
+import java.io.DataInput;
+import java.io.DataOutput;
+import java.io.IOException;
+import java.util.List;
+import java.util.Map;
+
+import org.apache.logging.log4j.Logger;
+
+import org.apache.geode.DataSerializer;
+import org.apache.geode.distributed.internal.ClusterDistributionManager;
+import org.apache.geode.distributed.internal.PooledDistributionMessage;
+import org.apache.geode.internal.cache.BucketRegionQueue;
+import org.apache.geode.internal.cache.InternalCache;
+import org.apache.geode.internal.cache.LocalRegion;
+import org.apache.geode.internal.cache.LocalRegion.InitializationLevel;
+import org.apache.geode.internal.cache.PartitionedRegion;
+import org.apache.geode.internal.cache.PartitionedRegionHelper;
+import org.apache.geode.internal.cache.wan.AbstractGatewaySender;
+import org.apache.geode.internal.serialization.DeserializationContext;
+import org.apache.geode.internal.serialization.SerializationContext;
+import org.apache.geode.logging.internal.log4j.api.LogService;
+
+/**
+ * Sets events in the remote secondary queues to possible duplicate
+ *
+ * @since Geode 1.15
+ */
+public class ParallelQueueSetPossibleDuplicateMessage extends PooledDistributionMessage {
+
+  private static final Logger logger = LogService.getLogger();
+
+  private int reason;
+  private Map regionToDispatchedKeysMap;
+
+  public static final int UNSUCCESSFULLY_DISPATCHED = 0;
+  public static final int RESET_BATCH = 1;
+  public static final int LOAD_FROM_TEMP_QUEUE = 2;
+  public static final int STOPPED_GATEWAY_SENDER = 3;
+
+
+  public ParallelQueueSetPossibleDuplicateMessage() {}
+
+  public ParallelQueueSetPossibleDuplicateMessage(int reason, Map rgnToDispatchedKeysMap) {
+    this.reason = reason;
+    this.regionToDispatchedKeysMap = rgnToDispatchedKeysMap;
+  }
+
+  @Override
+  public int getDSFID() {
+    return PARALLEL_QUEUE_SET_POSSIBLE_DUPLICATE_MESSAGE;
+  }
+
+  @Override
+  public String toString() {
+    String cname = getShortClassName();
+    final StringBuilder sb = new StringBuilder(cname);
+    sb.append("reason=" + reason);
+    sb.append(" regionToDispatchedKeysMap=" + regionToDispatchedKeysMap);
+    sb.append(" sender=").append(getSender());
+    return sb.toString();
+  }
+
+  @Override
+  protected void process(ClusterDistributionManager dm) {
+    final boolean isDebugEnabled = logger.isDebugEnabled();
+    final InternalCache cache = dm.getCache();
+
+    if (cache != null) {
+      final InitializationLevel oldLevel =
+          LocalRegion.setThreadInitLevelRequirement(BEFORE_INITIAL_IMAGE);
+      try {
+        for (Object name : regionToDispatchedKeysMap.keySet()) {
+          final String regionName = (String) name;
+          final PartitionedRegion region = (PartitionedRegion) cache.getRegion(regionName);
+          if (region == null) {
+            continue;
+          } else {
+            AbstractGatewaySender abstractSender = region.getParallelGatewaySender();
+            // Find the map: bucketId to dispatchedKeys
+            // Find the bucket
+            // Destroy the keys
+            Map bucketIdToDispatchedKeys = (Map) this.regionToDispatchedKeysMap.get(regionName);
+            for (Object bId : bucketIdToDispatchedKeys.keySet()) {
+              final String bucketFullPath =
+                  SEPARATOR + PartitionedRegionHelper.PR_ROOT_REGION_NAME + SEPARATOR
+                      + region.getBucketName((Integer) bId);
+              BucketRegionQueue brq =
+                  (BucketRegionQueue) cache.getInternalRegionByPath(bucketFullPath);
+
+              if (brq != null) {
+                if (reason == STOPPED_GATEWAY_SENDER) {
+                  brq.setReceivedGatewaySenderStoppedMessage(true);
+                }
+              }
+
+              if (isDebugEnabled) {
+                logger.debug(
+                    "ParallelQueueSetPossibleDuplicateMessage : The bucket in the cache is bucketRegionName : {} bucket: {}",
+                    bucketFullPath, brq);
+              }
+
+              List dispatchedKeys = (List) bucketIdToDispatchedKeys.get((Integer) bId);
+              if (dispatchedKeys != null && !dispatchedKeys.isEmpty()) {

Review Comment:
   This seems to suggest the insertion of `emptyList()` into the map previously commented was not necessary.
   .



##########
geode-core/src/main/java/org/apache/geode/internal/cache/wan/parallel/ParallelQueueSetPossibleDuplicateMessage.java:
##########
@@ -0,0 +1,166 @@
+/*
+ * 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.geode.internal.cache.wan.parallel;
+
+import static org.apache.geode.cache.Region.SEPARATOR;
+import static org.apache.geode.internal.cache.LocalRegion.InitializationLevel.BEFORE_INITIAL_IMAGE;
+
+import java.io.DataInput;
+import java.io.DataOutput;
+import java.io.IOException;
+import java.util.List;
+import java.util.Map;
+
+import org.apache.logging.log4j.Logger;
+
+import org.apache.geode.DataSerializer;
+import org.apache.geode.distributed.internal.ClusterDistributionManager;
+import org.apache.geode.distributed.internal.PooledDistributionMessage;
+import org.apache.geode.internal.cache.BucketRegionQueue;
+import org.apache.geode.internal.cache.InternalCache;
+import org.apache.geode.internal.cache.LocalRegion;
+import org.apache.geode.internal.cache.LocalRegion.InitializationLevel;
+import org.apache.geode.internal.cache.PartitionedRegion;
+import org.apache.geode.internal.cache.PartitionedRegionHelper;
+import org.apache.geode.internal.cache.wan.AbstractGatewaySender;
+import org.apache.geode.internal.serialization.DeserializationContext;
+import org.apache.geode.internal.serialization.SerializationContext;
+import org.apache.geode.logging.internal.log4j.api.LogService;
+
+/**
+ * Sets events in the remote secondary queues to possible duplicate
+ *
+ * @since Geode 1.15
+ */
+public class ParallelQueueSetPossibleDuplicateMessage extends PooledDistributionMessage {
+
+  private static final Logger logger = LogService.getLogger();
+
+  private int reason;
+  private Map regionToDispatchedKeysMap;
+
+  public static final int UNSUCCESSFULLY_DISPATCHED = 0;
+  public static final int RESET_BATCH = 1;
+  public static final int LOAD_FROM_TEMP_QUEUE = 2;
+  public static final int STOPPED_GATEWAY_SENDER = 3;
+
+
+  public ParallelQueueSetPossibleDuplicateMessage() {}
+
+  public ParallelQueueSetPossibleDuplicateMessage(int reason, Map rgnToDispatchedKeysMap) {
+    this.reason = reason;
+    this.regionToDispatchedKeysMap = rgnToDispatchedKeysMap;
+  }
+
+  @Override
+  public int getDSFID() {
+    return PARALLEL_QUEUE_SET_POSSIBLE_DUPLICATE_MESSAGE;
+  }
+
+  @Override
+  public String toString() {
+    String cname = getShortClassName();
+    final StringBuilder sb = new StringBuilder(cname);
+    sb.append("reason=" + reason);
+    sb.append(" regionToDispatchedKeysMap=" + regionToDispatchedKeysMap);
+    sb.append(" sender=").append(getSender());
+    return sb.toString();
+  }
+
+  @Override
+  protected void process(ClusterDistributionManager dm) {
+    final boolean isDebugEnabled = logger.isDebugEnabled();
+    final InternalCache cache = dm.getCache();
+
+    if (cache != null) {
+      final InitializationLevel oldLevel =
+          LocalRegion.setThreadInitLevelRequirement(BEFORE_INITIAL_IMAGE);
+      try {
+        for (Object name : regionToDispatchedKeysMap.keySet()) {
+          final String regionName = (String) name;
+          final PartitionedRegion region = (PartitionedRegion) cache.getRegion(regionName);
+          if (region == null) {
+            continue;
+          } else {

Review Comment:
   You an drop the else block to reduce the nesting. 



##########
geode-core/src/main/java/org/apache/geode/internal/cache/wan/parallel/ParallelQueueSetPossibleDuplicateMessage.java:
##########
@@ -0,0 +1,166 @@
+/*
+ * 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.geode.internal.cache.wan.parallel;
+
+import static org.apache.geode.cache.Region.SEPARATOR;
+import static org.apache.geode.internal.cache.LocalRegion.InitializationLevel.BEFORE_INITIAL_IMAGE;
+
+import java.io.DataInput;
+import java.io.DataOutput;
+import java.io.IOException;
+import java.util.List;
+import java.util.Map;
+
+import org.apache.logging.log4j.Logger;
+
+import org.apache.geode.DataSerializer;
+import org.apache.geode.distributed.internal.ClusterDistributionManager;
+import org.apache.geode.distributed.internal.PooledDistributionMessage;
+import org.apache.geode.internal.cache.BucketRegionQueue;
+import org.apache.geode.internal.cache.InternalCache;
+import org.apache.geode.internal.cache.LocalRegion;
+import org.apache.geode.internal.cache.LocalRegion.InitializationLevel;
+import org.apache.geode.internal.cache.PartitionedRegion;
+import org.apache.geode.internal.cache.PartitionedRegionHelper;
+import org.apache.geode.internal.cache.wan.AbstractGatewaySender;
+import org.apache.geode.internal.serialization.DeserializationContext;
+import org.apache.geode.internal.serialization.SerializationContext;
+import org.apache.geode.logging.internal.log4j.api.LogService;
+
+/**
+ * Sets events in the remote secondary queues to possible duplicate
+ *
+ * @since Geode 1.15
+ */
+public class ParallelQueueSetPossibleDuplicateMessage extends PooledDistributionMessage {
+
+  private static final Logger logger = LogService.getLogger();
+
+  private int reason;
+  private Map regionToDispatchedKeysMap;
+
+  public static final int UNSUCCESSFULLY_DISPATCHED = 0;
+  public static final int RESET_BATCH = 1;
+  public static final int LOAD_FROM_TEMP_QUEUE = 2;
+  public static final int STOPPED_GATEWAY_SENDER = 3;
+
+
+  public ParallelQueueSetPossibleDuplicateMessage() {}
+
+  public ParallelQueueSetPossibleDuplicateMessage(int reason, Map rgnToDispatchedKeysMap) {
+    this.reason = reason;
+    this.regionToDispatchedKeysMap = rgnToDispatchedKeysMap;
+  }
+
+  @Override
+  public int getDSFID() {
+    return PARALLEL_QUEUE_SET_POSSIBLE_DUPLICATE_MESSAGE;
+  }
+
+  @Override
+  public String toString() {
+    String cname = getShortClassName();
+    final StringBuilder sb = new StringBuilder(cname);
+    sb.append("reason=" + reason);
+    sb.append(" regionToDispatchedKeysMap=" + regionToDispatchedKeysMap);
+    sb.append(" sender=").append(getSender());
+    return sb.toString();
+  }
+
+  @Override
+  protected void process(ClusterDistributionManager dm) {
+    final boolean isDebugEnabled = logger.isDebugEnabled();
+    final InternalCache cache = dm.getCache();
+
+    if (cache != null) {
+      final InitializationLevel oldLevel =
+          LocalRegion.setThreadInitLevelRequirement(BEFORE_INITIAL_IMAGE);
+      try {
+        for (Object name : regionToDispatchedKeysMap.keySet()) {
+          final String regionName = (String) name;
+          final PartitionedRegion region = (PartitionedRegion) cache.getRegion(regionName);
+          if (region == null) {
+            continue;
+          } else {
+            AbstractGatewaySender abstractSender = region.getParallelGatewaySender();
+            // Find the map: bucketId to dispatchedKeys
+            // Find the bucket
+            // Destroy the keys
+            Map bucketIdToDispatchedKeys = (Map) this.regionToDispatchedKeysMap.get(regionName);
+            for (Object bId : bucketIdToDispatchedKeys.keySet()) {
+              final String bucketFullPath =
+                  SEPARATOR + PartitionedRegionHelper.PR_ROOT_REGION_NAME + SEPARATOR
+                      + region.getBucketName((Integer) bId);
+              BucketRegionQueue brq =
+                  (BucketRegionQueue) cache.getInternalRegionByPath(bucketFullPath);
+
+              if (brq != null) {
+                if (reason == STOPPED_GATEWAY_SENDER) {
+                  brq.setReceivedGatewaySenderStoppedMessage(true);
+                }
+              }
+
+              if (isDebugEnabled) {
+                logger.debug(
+                    "ParallelQueueSetPossibleDuplicateMessage : The bucket in the cache is bucketRegionName : {} bucket: {}",
+                    bucketFullPath, brq);
+              }
+
+              List dispatchedKeys = (List) bucketIdToDispatchedKeys.get((Integer) bId);
+              if (dispatchedKeys != null && !dispatchedKeys.isEmpty()) {
+                for (Object key : dispatchedKeys) {
+                  // First, clear the Event from tempQueueEvents at AbstractGatewaySender level, if
+                  // exists

Review Comment:
   Formatting?



-- 
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@geode.apache.org

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