You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@activemq.apache.org by cl...@apache.org on 2019/08/15 18:06:43 UTC

[activemq-artemis] branch master updated: ARTEMIS-2453 Fixing deadLock between destroyQueue and removeAddressInfo

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

clebertsuconic pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/activemq-artemis.git


The following commit(s) were added to refs/heads/master by this push:
     new 144c21f  ARTEMIS-2453 Fixing deadLock between destroyQueue and removeAddressInfo
     new 349fea8  This closes #2799
144c21f is described below

commit 144c21fb6f9489174fc8ee0e8a37459a657b2b7a
Author: Clebert Suconic <cl...@apache.org>
AuthorDate: Thu Aug 15 11:23:25 2019 -0400

    ARTEMIS-2453 Fixing deadLock between destroyQueue and removeAddressInfo
---
 .../core/postoffice/impl/PostOfficeImpl.java       |  14 +--
 .../addressing/TwoWaysRemoveAddressTest.java       | 101 +++++++++++++++++++++
 2 files changed, 106 insertions(+), 9 deletions(-)

diff --git a/artemis-server/src/main/java/org/apache/activemq/artemis/core/postoffice/impl/PostOfficeImpl.java b/artemis-server/src/main/java/org/apache/activemq/artemis/core/postoffice/impl/PostOfficeImpl.java
index c7559be..7156514 100644
--- a/artemis-server/src/main/java/org/apache/activemq/artemis/core/postoffice/impl/PostOfficeImpl.java
+++ b/artemis-server/src/main/java/org/apache/activemq/artemis/core/postoffice/impl/PostOfficeImpl.java
@@ -137,8 +137,6 @@ public class PostOfficeImpl implements PostOffice, NotificationListener, Binding
 
    private final ActiveMQServer server;
 
-   private final Object addressLock = new Object();
-
    public PostOfficeImpl(final ActiveMQServer server,
                          final StorageManager storageManager,
                          final PagingManager pagingManager,
@@ -441,7 +439,7 @@ public class PostOfficeImpl implements PostOffice, NotificationListener, Binding
    }
 
    private boolean internalAddressInfo(AddressInfo addressInfo, boolean reload) throws Exception {
-      synchronized (addressLock) {
+      synchronized (this) {
          if (server.hasBrokerAddressPlugins()) {
             server.callBrokerAddressPlugins(plugin -> plugin.beforeAddAddress(addressInfo, reload));
          }
@@ -484,7 +482,7 @@ public class PostOfficeImpl implements PostOffice, NotificationListener, Binding
                                    Long delayBeforeDispatch,
                                    SimpleString user,
                                    Boolean configurationManaged) throws Exception {
-      synchronized (addressLock) {
+      synchronized (this) {
          final QueueBinding queueBinding = (QueueBinding) addressManager.getBinding(name);
          if (queueBinding == null) {
             return null;
@@ -597,7 +595,7 @@ public class PostOfficeImpl implements PostOffice, NotificationListener, Binding
    @Override
    public AddressInfo updateAddressInfo(SimpleString addressName,
                                         EnumSet<RoutingType> routingTypes) throws Exception {
-      synchronized (addressLock) {
+      synchronized (this) {
          if (server.hasBrokerAddressPlugins()) {
             server.callBrokerAddressPlugins(plugin -> plugin.beforeUpdateAddress(addressName, routingTypes));
          }
@@ -619,7 +617,7 @@ public class PostOfficeImpl implements PostOffice, NotificationListener, Binding
 
    @Override
    public AddressInfo removeAddressInfo(SimpleString address, boolean force) throws Exception {
-      synchronized (addressLock) {
+      synchronized (this) {
          if (server.hasBrokerAddressPlugins()) {
             server.callBrokerAddressPlugins(plugin -> plugin.beforeRemoveAddress(address));
          }
@@ -649,9 +647,7 @@ public class PostOfficeImpl implements PostOffice, NotificationListener, Binding
 
    @Override
    public AddressInfo getAddressInfo(SimpleString addressName) {
-      synchronized (addressLock) {
-         return addressManager.getAddressInfo(addressName);
-      }
+      return addressManager.getAddressInfo(addressName);
    }
 
    @Override
diff --git a/tests/integration-tests/src/test/java/org/apache/activemq/artemis/tests/integration/addressing/TwoWaysRemoveAddressTest.java b/tests/integration-tests/src/test/java/org/apache/activemq/artemis/tests/integration/addressing/TwoWaysRemoveAddressTest.java
new file mode 100644
index 0000000..5ffe42e
--- /dev/null
+++ b/tests/integration-tests/src/test/java/org/apache/activemq/artemis/tests/integration/addressing/TwoWaysRemoveAddressTest.java
@@ -0,0 +1,101 @@
+/*
+ * 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.activemq.artemis.tests.integration.addressing;
+
+import java.util.concurrent.CyclicBarrier;
+import java.util.concurrent.TimeUnit;
+import java.util.concurrent.atomic.AtomicInteger;
+
+import org.apache.activemq.artemis.api.core.RoutingType;
+import org.apache.activemq.artemis.api.core.SimpleString;
+import org.apache.activemq.artemis.core.server.ActiveMQServer;
+import org.apache.activemq.artemis.tests.util.ActiveMQTestBase;
+import org.jboss.logging.Logger;
+import org.junit.Assert;
+import org.junit.Test;
+
+/**
+ * This test is simulating a dead lock that may happen while removing addresses.
+ */
+public class TwoWaysRemoveAddressTest extends ActiveMQTestBase {
+
+   private static Logger logger = Logger.getLogger(TwoWaysRemoveAddressTest.class);
+
+   @Test(timeout = 60_000)
+   public void testDeadLock() throws Throwable  {
+      ActiveMQServer server = addServer(createServer(false));
+      server.start();
+
+      final int retries = 10;
+      CyclicBarrier barrier = new CyclicBarrier(2);
+
+      AtomicInteger errors = new AtomicInteger(0);
+
+      Thread createAndDestroy1 = new Thread() {
+         @Override
+         public void run() {
+
+            try {
+               barrier.await(10, TimeUnit.SECONDS);
+
+               for (int i = 0; i < retries; i++) {
+                  logger.debug("Removed queue on thread 1 ::" + i);
+                  server.createQueue(SimpleString.toSimpleString("address_1_" + i), RoutingType.ANYCAST, SimpleString.toSimpleString("queueName_1_" + i), null, true, false);
+                  server.destroyQueue(SimpleString.toSimpleString("queueName_1_" + i));
+               }
+            } catch (Throwable e) {
+               logger.warn(e.getMessage(), e);
+               errors.incrementAndGet();
+            }
+         }
+      };
+
+      Thread createAndDestroy2 = new Thread() {
+         @Override
+         public void run() {
+
+            try {
+               barrier.await(10, TimeUnit.SECONDS);
+
+               for (int i = 0; i < retries; i++) {
+                  logger.debug("Removed queue on thread 2 ::" + i);
+                  server.createQueue(SimpleString.toSimpleString("address_2_" + i), RoutingType.ANYCAST, SimpleString.toSimpleString("queueName_2_" + i), null, true, false);
+                  server.removeAddressInfo(SimpleString.toSimpleString("address_2_" + i), null, true);
+               }
+            } catch (Throwable e) {
+               logger.warn(e.getMessage(), e);
+               errors.incrementAndGet();
+            }
+         }
+      };
+
+      createAndDestroy1.start();
+      createAndDestroy2.start();
+
+      createAndDestroy1.join(10_000);
+      createAndDestroy2.join(10_000);
+
+      Assert.assertFalse(createAndDestroy1.isAlive());
+      Assert.assertFalse(createAndDestroy2.isAlive());
+
+
+      Assert.assertEquals(0, errors.get());
+
+   }
+
+}