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 2018/01/31 23:19:05 UTC

[1/2] activemq-artemis git commit: ARTEMIS-1644 legacy clients can't access resources with old prefixes

Repository: activemq-artemis
Updated Branches:
  refs/heads/master 3d79a0896 -> 5c6f0c6df


ARTEMIS-1644 legacy clients can't access resources with old prefixes


Project: http://git-wip-us.apache.org/repos/asf/activemq-artemis/repo
Commit: http://git-wip-us.apache.org/repos/asf/activemq-artemis/commit/9c8bf2f2
Tree: http://git-wip-us.apache.org/repos/asf/activemq-artemis/tree/9c8bf2f2
Diff: http://git-wip-us.apache.org/repos/asf/activemq-artemis/diff/9c8bf2f2

Branch: refs/heads/master
Commit: 9c8bf2f2cac07bc56521fc007f005e0c955d1624
Parents: 3d79a08
Author: Justin Bertram <jb...@apache.org>
Authored: Tue Jan 30 13:04:51 2018 -0600
Committer: Clebert Suconic <cl...@apache.org>
Committed: Wed Jan 31 18:13:29 2018 -0500

----------------------------------------------------------------------
 .../artemis/cli/commands/etc/broker.xml         |   4 +
 .../cli/commands/etc/hornetq-acceptor.txt       |   2 +-
 .../impl/wireformat/QueueAbstractPacket.java    |  81 ++++++--------
 .../artemis/api/core/QueueAbstractTest.java     |  48 ---------
 .../hornetq/HornetQProtocolManager.java         |   5 +
 .../hornetq/HornetQProtocolManagerFactory.java  |   5 +-
 .../core/ServerSessionPacketHandler.java        |   8 +-
 .../core/impl/ActiveMQPacketHandler.java        |   9 +-
 .../protocol/core/impl/CoreProtocolManager.java |   2 +-
 .../addressConfig/artemisServer.groovy          |   2 +-
 .../resources/exportimport/artemisServer.groovy |  60 +++++++++++
 .../prefixSendAckTest/artemisServer.groovy      |  60 +++++++++++
 .../prefixSendAckTest/sendAckMessages.groovy    | 105 +++++++++++++++++++
 .../main/resources/servers/artemisServer.groovy |   2 +-
 .../tests/compatibility/ExportImportTest.java   |   8 ++
 .../tests/compatibility/PrefixSendAckTest.java  |  59 +++++++++++
 .../tests/compatibility/VersionedBaseTest.java  |   8 +-
 17 files changed, 351 insertions(+), 117 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/activemq-artemis/blob/9c8bf2f2/artemis-cli/src/main/resources/org/apache/activemq/artemis/cli/commands/etc/broker.xml
----------------------------------------------------------------------
diff --git a/artemis-cli/src/main/resources/org/apache/activemq/artemis/cli/commands/etc/broker.xml b/artemis-cli/src/main/resources/org/apache/activemq/artemis/cli/commands/etc/broker.xml
index ec591f3..8e90625 100644
--- a/artemis-cli/src/main/resources/org/apache/activemq/artemis/cli/commands/etc/broker.xml
+++ b/artemis-cli/src/main/resources/org/apache/activemq/artemis/cli/commands/etc/broker.xml
@@ -77,6 +77,10 @@ ${global-max-section}
          <!-- amqpCredits: The number of credits sent to AMQP producers -->
          <!-- amqpLowCredits: The server will send the # credits specified at amqpCredits at this low mark -->
 
+         <!-- Note: If an acceptor needs to be compatible with HornetQ and/or Artemis 1.x clients add
+                    "anycastPrefix=jms.queue.;multicastPrefix=jms.topic." to the acceptor url.
+                    See https://issues.apache.org/jira/browse/ARTEMIS-1644 for more information. -->
+
          <!-- Acceptor for every supported protocol -->
          <acceptor name="artemis">tcp://${host}:${default.port}?tcpSendBufferSize=1048576;tcpReceiveBufferSize=1048576;protocols=CORE,AMQP,STOMP,HORNETQ,MQTT,OPENWIRE;useEpoll=true;amqpCredits=1000;amqpLowCredits=300</acceptor>
 ${amqp-acceptor}${stomp-acceptor}${hornetq-acceptor}${mqtt-acceptor}

http://git-wip-us.apache.org/repos/asf/activemq-artemis/blob/9c8bf2f2/artemis-cli/src/main/resources/org/apache/activemq/artemis/cli/commands/etc/hornetq-acceptor.txt
----------------------------------------------------------------------
diff --git a/artemis-cli/src/main/resources/org/apache/activemq/artemis/cli/commands/etc/hornetq-acceptor.txt b/artemis-cli/src/main/resources/org/apache/activemq/artemis/cli/commands/etc/hornetq-acceptor.txt
index da77ebe..c90e011 100644
--- a/artemis-cli/src/main/resources/org/apache/activemq/artemis/cli/commands/etc/hornetq-acceptor.txt
+++ b/artemis-cli/src/main/resources/org/apache/activemq/artemis/cli/commands/etc/hornetq-acceptor.txt
@@ -1,3 +1,3 @@
 
          <!-- HornetQ Compatibility Acceptor.  Enables HornetQ Core and STOMP for legacy HornetQ clients. -->
-         <acceptor name="hornetq">tcp://${host}:${hq.port}?protocols=HORNETQ,STOMP;useEpoll=true</acceptor>
+         <acceptor name="hornetq">tcp://${host}:${hq.port}?anycastPrefix=jms.queue.;multicastPrefix=jms.topic.;protocols=HORNETQ,STOMP;useEpoll=true</acceptor>

http://git-wip-us.apache.org/repos/asf/activemq-artemis/blob/9c8bf2f2/artemis-core-client/src/main/java/org/apache/activemq/artemis/core/protocol/core/impl/wireformat/QueueAbstractPacket.java
----------------------------------------------------------------------
diff --git a/artemis-core-client/src/main/java/org/apache/activemq/artemis/core/protocol/core/impl/wireformat/QueueAbstractPacket.java b/artemis-core-client/src/main/java/org/apache/activemq/artemis/core/protocol/core/impl/wireformat/QueueAbstractPacket.java
index 641d7cc..0e07c60 100644
--- a/artemis-core-client/src/main/java/org/apache/activemq/artemis/core/protocol/core/impl/wireformat/QueueAbstractPacket.java
+++ b/artemis-core-client/src/main/java/org/apache/activemq/artemis/core/protocol/core/impl/wireformat/QueueAbstractPacket.java
@@ -28,35 +28,15 @@ import org.apache.activemq.artemis.core.protocol.core.impl.PacketImpl;
 public abstract class QueueAbstractPacket extends PacketImpl {
 
    protected SimpleString queueName;
-   protected SimpleString oldVersionQueueName;
 
    protected SimpleString address;
-   protected SimpleString oldVersionAddresseName;
 
-   public SimpleString getQueueName(int clientVersion) {
-
-      if (clientVersion < ADDRESSING_CHANGE_VERSION) {
-         if (oldVersionQueueName == null) {
-            oldVersionQueueName = convertName(queueName);
-         }
-
-         return oldVersionQueueName;
-      } else {
-         return queueName;
-      }
+   public SimpleString getQueueName() {
+      return queueName;
    }
 
-   public SimpleString getAddress(int clientVersion) {
-
-      if (clientVersion < ADDRESSING_CHANGE_VERSION) {
-         if (oldVersionAddresseName == null) {
-            oldVersionAddresseName = convertName(address);
-         }
-
-         return oldVersionAddresseName;
-      } else {
-         return address;
-      }
+   public SimpleString getAddress() {
+      return address;
    }
 
    /**
@@ -69,30 +49,29 @@ public abstract class QueueAbstractPacket extends PacketImpl {
     */
    public final List<SimpleString> convertQueueNames(int clientVersion, List<SimpleString> queueNames) {
       if (clientVersion < ADDRESSING_CHANGE_VERSION) {
-         return applyAddressPrefixTo(queueNames);
-      } else {
-         return queueNames;
-      }
-   }
-
-   private List<SimpleString> applyAddressPrefixTo(List<SimpleString> queueNames) {
-      final int names = queueNames.size();
-      if (names == 0) {
-         return Collections.emptyList();
-      } else {
-         final SimpleString address = this.address;
-         final SimpleString prefix = jmsPrefixOf(address);
-         if (prefix != null) {
-            final List<SimpleString> prefixedQueueNames = new ArrayList<>(names);
-            for (int i = 0; i < names; i++) {
-               final SimpleString oldQueueNames = queueNames.get(i);
-               final SimpleString prefixedQueueName = prefix.concat(oldQueueNames);
-               prefixedQueueNames.add(prefixedQueueName);
-            }
-            return prefixedQueueNames;
+         final int names = queueNames.size();
+         if (names == 0) {
+            return Collections.emptyList();
          } else {
-            return queueNames;
+            final SimpleString prefix = jmsPrefixOf(this.address);
+            if (prefix != null) {
+               final List<SimpleString> prefixedQueueNames = new ArrayList<>(names);
+               for (int i = 0; i < names; i++) {
+                  final SimpleString oldQueueName = queueNames.get(i);
+                  if (oldQueueName.startsWith(prefix)) {
+                     prefixedQueueNames.add(oldQueueName);
+                  } else {
+                     final SimpleString prefixedQueueName = prefix.concat(oldQueueName);
+                     prefixedQueueNames.add(prefixedQueueName);
+                  }
+               }
+               return prefixedQueueNames;
+            } else {
+               return queueNames;
+            }
          }
+      } else {
+         return queueNames;
       }
    }
 
@@ -111,10 +90,12 @@ public abstract class QueueAbstractPacket extends PacketImpl {
    }
 
    public static SimpleString getOldPrefixedAddress(SimpleString address, RoutingType routingType) {
-      switch (routingType) {
-         case MULTICAST: return OLD_TOPIC_PREFIX.concat(address);
-         case ANYCAST: return OLD_QUEUE_PREFIX.concat(address);
-         default: return address;
+      if (routingType == RoutingType.MULTICAST && !address.startsWith(OLD_TOPIC_PREFIX)) {
+         return OLD_TOPIC_PREFIX.concat(address);
+      } else if (routingType == RoutingType.ANYCAST && !address.startsWith(OLD_QUEUE_PREFIX)) {
+         return OLD_QUEUE_PREFIX.concat(address);
       }
+
+      return address;
    }
 }

http://git-wip-us.apache.org/repos/asf/activemq-artemis/blob/9c8bf2f2/artemis-core-client/src/test/java/org/apache/activemq/artemis/api/core/QueueAbstractTest.java
----------------------------------------------------------------------
diff --git a/artemis-core-client/src/test/java/org/apache/activemq/artemis/api/core/QueueAbstractTest.java b/artemis-core-client/src/test/java/org/apache/activemq/artemis/api/core/QueueAbstractTest.java
deleted file mode 100644
index e75c184..0000000
--- a/artemis-core-client/src/test/java/org/apache/activemq/artemis/api/core/QueueAbstractTest.java
+++ /dev/null
@@ -1,48 +0,0 @@
-/**
- * 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.api.core;
-
-import org.apache.activemq.artemis.core.protocol.core.impl.wireformat.QueueAbstractPacket;
-import org.junit.Assert;
-import org.junit.Test;
-
-public class QueueAbstractTest {
-
-   class MyTest extends QueueAbstractPacket {
-
-      MyTest(String name) {
-         super((byte)0);
-         this.queueName = SimpleString.toSimpleString(name);
-      }
-   }
-
-
-   @Test
-   public void testOldTopic() {
-      MyTest test = new MyTest("jms.topic.mytopic");
-
-      Assert.assertEquals("mytopic", test.getQueueName(127).toString());
-   }
-
-   @Test
-   public void testOldQueue() {
-      MyTest test = new MyTest("jms.queue.myQueue");
-
-      Assert.assertEquals("myQueue", test.getQueueName(127).toString());
-   }
-}

http://git-wip-us.apache.org/repos/asf/activemq-artemis/blob/9c8bf2f2/artemis-protocols/artemis-hornetq-protocol/src/main/java/org/apache/activemq/artemis/core/protocol/hornetq/HornetQProtocolManager.java
----------------------------------------------------------------------
diff --git a/artemis-protocols/artemis-hornetq-protocol/src/main/java/org/apache/activemq/artemis/core/protocol/hornetq/HornetQProtocolManager.java b/artemis-protocols/artemis-hornetq-protocol/src/main/java/org/apache/activemq/artemis/core/protocol/hornetq/HornetQProtocolManager.java
index de7d2ff..df51cb0 100644
--- a/artemis-protocols/artemis-hornetq-protocol/src/main/java/org/apache/activemq/artemis/core/protocol/hornetq/HornetQProtocolManager.java
+++ b/artemis-protocols/artemis-hornetq-protocol/src/main/java/org/apache/activemq/artemis/core/protocol/hornetq/HornetQProtocolManager.java
@@ -63,4 +63,9 @@ class HornetQProtocolManager extends CoreProtocolManager {
       String frameStart = new String(array, StandardCharsets.US_ASCII);
       return frameStart.startsWith("HORNETQ");
    }
+
+   @Override
+   public String toString() {
+      return "HornetQProtocolManager(server=" + super.server + ")";
+   }
 }

http://git-wip-us.apache.org/repos/asf/activemq-artemis/blob/9c8bf2f2/artemis-protocols/artemis-hornetq-protocol/src/main/java/org/apache/activemq/artemis/core/protocol/hornetq/HornetQProtocolManagerFactory.java
----------------------------------------------------------------------
diff --git a/artemis-protocols/artemis-hornetq-protocol/src/main/java/org/apache/activemq/artemis/core/protocol/hornetq/HornetQProtocolManagerFactory.java b/artemis-protocols/artemis-hornetq-protocol/src/main/java/org/apache/activemq/artemis/core/protocol/hornetq/HornetQProtocolManagerFactory.java
index 9fd851f..fb9523d 100644
--- a/artemis-protocols/artemis-hornetq-protocol/src/main/java/org/apache/activemq/artemis/core/protocol/hornetq/HornetQProtocolManagerFactory.java
+++ b/artemis-protocols/artemis-hornetq-protocol/src/main/java/org/apache/activemq/artemis/core/protocol/hornetq/HornetQProtocolManagerFactory.java
@@ -25,6 +25,7 @@ import org.apache.activemq.artemis.core.protocol.core.impl.CoreProtocolManagerFa
 import org.apache.activemq.artemis.core.server.ActiveMQServer;
 import org.apache.activemq.artemis.spi.core.protocol.ProtocolManager;
 import org.apache.activemq.artemis.spi.core.protocol.ProtocolManagerFactory;
+import org.apache.activemq.artemis.utils.uri.BeanSupport;
 import org.osgi.service.component.annotations.Component;
 
 @Component(service = ProtocolManagerFactory.class)
@@ -40,7 +41,7 @@ public class HornetQProtocolManagerFactory extends CoreProtocolManagerFactory {
    public ProtocolManager createProtocolManager(final ActiveMQServer server,
                                                 final Map<String, Object> parameters,
                                                 final List<BaseInterceptor> incomingInterceptors,
-                                                List<BaseInterceptor> outgoingInterceptors) {
+                                                List<BaseInterceptor> outgoingInterceptors) throws Exception {
 
       List<Interceptor> hqIncoming = filterInterceptors(incomingInterceptors);
       List<Interceptor> hqOutgoing = filterInterceptors(outgoingInterceptors);
@@ -48,7 +49,7 @@ public class HornetQProtocolManagerFactory extends CoreProtocolManagerFactory {
       hqIncoming.add(new HQPropertiesConversionInterceptor(true));
       hqOutgoing.add(new HQPropertiesConversionInterceptor(false));
 
-      return new HornetQProtocolManager(this, server, hqIncoming, hqOutgoing);
+      return BeanSupport.setData(new HornetQProtocolManager(this, server, hqIncoming, hqOutgoing), parameters);
    }
 
    @Override

http://git-wip-us.apache.org/repos/asf/activemq-artemis/blob/9c8bf2f2/artemis-server/src/main/java/org/apache/activemq/artemis/core/protocol/core/ServerSessionPacketHandler.java
----------------------------------------------------------------------
diff --git a/artemis-server/src/main/java/org/apache/activemq/artemis/core/protocol/core/ServerSessionPacketHandler.java b/artemis-server/src/main/java/org/apache/activemq/artemis/core/protocol/core/ServerSessionPacketHandler.java
index e93dd94..d47fbff 100644
--- a/artemis-server/src/main/java/org/apache/activemq/artemis/core/protocol/core/ServerSessionPacketHandler.java
+++ b/artemis-server/src/main/java/org/apache/activemq/artemis/core/protocol/core/ServerSessionPacketHandler.java
@@ -314,11 +314,11 @@ public class ServerSessionPacketHandler implements ChannelHandler {
                case SESS_CREATECONSUMER: {
                   SessionCreateConsumerMessage request = (SessionCreateConsumerMessage) packet;
                   requiresResponse = request.isRequiresResponse();
-                  session.createConsumer(request.getID(), request.getQueueName(remotingConnection.getChannelVersion()), request.getFilterString(), request.isBrowseOnly());
+                  session.createConsumer(request.getID(), request.getQueueName(), request.getFilterString(), request.isBrowseOnly());
                   if (requiresResponse) {
                      // We send back queue information on the queue as a response- this allows the queue to
                      // be automatically recreated on failover
-                     QueueQueryResult queueQueryResult = session.executeQueueQuery(request.getQueueName(remotingConnection.getChannelVersion()));
+                     QueueQueryResult queueQueryResult = session.executeQueueQuery(request.getQueueName());
 
                      if (channel.supports(PacketImpl.SESS_QUEUEQUERY_RESP_V3)) {
                         response = new SessionQueueQueryResponseMessage_V3(queueQueryResult);
@@ -387,7 +387,7 @@ public class ServerSessionPacketHandler implements ChannelHandler {
                case SESS_QUEUEQUERY: {
                   requiresResponse = true;
                   SessionQueueQueryMessage request = (SessionQueueQueryMessage) packet;
-                  QueueQueryResult result = session.executeQueueQuery(request.getQueueName(remotingConnection.getChannelVersion()));
+                  QueueQueryResult result = session.executeQueueQuery(request.getQueueName());
 
                   if (remotingConnection.getChannelVersion() < PacketImpl.ADDRESSING_CHANGE_VERSION) {
                      result.setAddress(SessionQueueQueryMessage.getOldPrefixedAddress(result.getAddress(), result.getRoutingType()));
@@ -406,7 +406,7 @@ public class ServerSessionPacketHandler implements ChannelHandler {
                   requiresResponse = true;
                   SessionBindingQueryMessage request = (SessionBindingQueryMessage) packet;
                   final int clientVersion = remotingConnection.getChannelVersion();
-                  BindingQueryResult result = session.executeBindingQuery(request.getAddress(clientVersion));
+                  BindingQueryResult result = session.executeBindingQuery(request.getAddress());
 
                   /* if the session is JMS and it's from an older client then we need to add the old prefix to the queue
                    * names otherwise the older client won't realize the queue exists and will try to create it and receive

http://git-wip-us.apache.org/repos/asf/activemq-artemis/blob/9c8bf2f2/artemis-server/src/main/java/org/apache/activemq/artemis/core/protocol/core/impl/ActiveMQPacketHandler.java
----------------------------------------------------------------------
diff --git a/artemis-server/src/main/java/org/apache/activemq/artemis/core/protocol/core/impl/ActiveMQPacketHandler.java b/artemis-server/src/main/java/org/apache/activemq/artemis/core/protocol/core/impl/ActiveMQPacketHandler.java
index d38483a..ebd1843 100644
--- a/artemis-server/src/main/java/org/apache/activemq/artemis/core/protocol/core/impl/ActiveMQPacketHandler.java
+++ b/artemis-server/src/main/java/org/apache/activemq/artemis/core/protocol/core/impl/ActiveMQPacketHandler.java
@@ -16,7 +16,6 @@
  */
 package org.apache.activemq.artemis.core.protocol.core.impl;
 
-import java.util.HashMap;
 import java.util.Map;
 
 import org.apache.activemq.artemis.api.core.ActiveMQClusterSecurityException;
@@ -24,6 +23,7 @@ import org.apache.activemq.artemis.api.core.ActiveMQException;
 import org.apache.activemq.artemis.api.core.ActiveMQExceptionType;
 import org.apache.activemq.artemis.api.core.ActiveMQInternalErrorException;
 import org.apache.activemq.artemis.api.core.ActiveMQSecurityException;
+import org.apache.activemq.artemis.api.core.RoutingType;
 import org.apache.activemq.artemis.api.core.SimpleString;
 import org.apache.activemq.artemis.core.persistence.OperationContext;
 import org.apache.activemq.artemis.core.protocol.core.Channel;
@@ -43,7 +43,6 @@ import org.apache.activemq.artemis.core.security.ActiveMQPrincipal;
 import org.apache.activemq.artemis.core.server.ActiveMQMessageBundle;
 import org.apache.activemq.artemis.core.server.ActiveMQServer;
 import org.apache.activemq.artemis.core.server.ActiveMQServerLogger;
-import org.apache.activemq.artemis.api.core.RoutingType;
 import org.apache.activemq.artemis.core.server.ServerProducer;
 import org.apache.activemq.artemis.core.server.ServerSession;
 import org.apache.activemq.artemis.core.server.impl.ServerProducerImpl;
@@ -163,12 +162,6 @@ public class ActiveMQPacketHandler implements ChannelHandler {
 
          Map<SimpleString, RoutingType> routingTypeMap = protocolManager.getPrefixes();
 
-         if (connection.getChannelVersion() < PacketImpl.ADDRESSING_CHANGE_VERSION) {
-            routingTypeMap = new HashMap<>();
-            routingTypeMap.put(PacketImpl.OLD_QUEUE_PREFIX, RoutingType.ANYCAST);
-            routingTypeMap.put(PacketImpl.OLD_TOPIC_PREFIX, RoutingType.MULTICAST);
-         }
-
          CoreSessionCallback sessionCallback = new CoreSessionCallback(request.getName(), protocolManager, channel, connection);
          ServerSession session = server.createSession(request.getName(), activeMQPrincipal == null ? request.getUsername() : activeMQPrincipal.getUserName(), activeMQPrincipal == null ? request.getPassword() : activeMQPrincipal.getPassword(), request.getMinLargeMessageSize(), connection, request.isAutoCommitSends(), request.isAutoCommitAcks(), request.isPreAcknowledge(), request.isXA(), request.getDefaultAddress(), sessionCallback, true, sessionOperationContext, routingTypeMap);
          ServerProducer serverProducer = new ServerProducerImpl(session.getName(), "CORE", request.getDefaultAddress());

http://git-wip-us.apache.org/repos/asf/activemq-artemis/blob/9c8bf2f2/artemis-server/src/main/java/org/apache/activemq/artemis/core/protocol/core/impl/CoreProtocolManager.java
----------------------------------------------------------------------
diff --git a/artemis-server/src/main/java/org/apache/activemq/artemis/core/protocol/core/impl/CoreProtocolManager.java b/artemis-server/src/main/java/org/apache/activemq/artemis/core/protocol/core/impl/CoreProtocolManager.java
index af9e131..0b5432a 100644
--- a/artemis-server/src/main/java/org/apache/activemq/artemis/core/protocol/core/impl/CoreProtocolManager.java
+++ b/artemis-server/src/main/java/org/apache/activemq/artemis/core/protocol/core/impl/CoreProtocolManager.java
@@ -68,7 +68,7 @@ public class CoreProtocolManager implements ProtocolManager<Interceptor> {
 
    private static final List<String> websocketRegistryNames = Collections.EMPTY_LIST;
 
-   private final ActiveMQServer server;
+   protected final ActiveMQServer server;
 
    private final List<Interceptor> incomingInterceptors;
 

http://git-wip-us.apache.org/repos/asf/activemq-artemis/blob/9c8bf2f2/tests/compatibility-tests/src/main/resources/addressConfig/artemisServer.groovy
----------------------------------------------------------------------
diff --git a/tests/compatibility-tests/src/main/resources/addressConfig/artemisServer.groovy b/tests/compatibility-tests/src/main/resources/addressConfig/artemisServer.groovy
index 854f542..31faf30 100644
--- a/tests/compatibility-tests/src/main/resources/addressConfig/artemisServer.groovy
+++ b/tests/compatibility-tests/src/main/resources/addressConfig/artemisServer.groovy
@@ -34,7 +34,7 @@ configuration = new ConfigurationImpl();
 configuration.setJournalType(JournalType.NIO);
 System.out.println("folder:: " + folder);
 configuration.setBrokerInstance(new File(folder + "/" + id));
-configuration.addAcceptorConfiguration("artemis", "tcp://0.0.0.0:61616");
+configuration.addAcceptorConfiguration("artemis", "tcp://0.0.0.0:61616?anycastPrefix=jms.queue.&multicastPrefix=jms.topic.");
 configuration.setSecurityEnabled(false);
 configuration.setPersistenceEnabled(false);
 configuration.addAddressesSetting("myQueue", new AddressSettings().setAddressFullMessagePolicy(AddressFullMessagePolicy.PAGE).setMaxSizeBytes(1024 * 1024 * 1024).setPageSizeBytes(1024));

http://git-wip-us.apache.org/repos/asf/activemq-artemis/blob/9c8bf2f2/tests/compatibility-tests/src/main/resources/exportimport/artemisServer.groovy
----------------------------------------------------------------------
diff --git a/tests/compatibility-tests/src/main/resources/exportimport/artemisServer.groovy b/tests/compatibility-tests/src/main/resources/exportimport/artemisServer.groovy
new file mode 100644
index 0000000..c6d7a7b
--- /dev/null
+++ b/tests/compatibility-tests/src/main/resources/exportimport/artemisServer.groovy
@@ -0,0 +1,60 @@
+package servers
+/*
+ * 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.
+ */
+
+// starts an artemis server
+
+import org.apache.activemq.artemis.core.config.impl.ConfigurationImpl;
+import org.apache.activemq.artemis.core.server.JournalType;
+import org.apache.activemq.artemis.core.settings.impl.AddressSettings;
+import org.apache.activemq.artemis.jms.server.config.impl.JMSConfigurationImpl;
+import org.apache.activemq.artemis.jms.server.embedded.EmbeddedJMS
+
+
+String folder = arg[0];
+String id = arg[1];
+String type = arg[2];
+String producer = arg[3];
+String consumer = arg[4];
+
+println("type = " + type);
+
+configuration = new ConfigurationImpl();
+configuration.setJournalType(JournalType.NIO);
+System.out.println("folder:: " + folder);
+configuration.setBrokerInstance(new File(folder + "/" + id));
+configuration.addAcceptorConfiguration("artemis", "tcp://0.0.0.0:61616");
+configuration.setSecurityEnabled(false);
+configuration.setPersistenceEnabled(persistent);
+try {
+    if (!type.startsWith("ARTEMIS-1")) {
+        configuration.addAddressesSetting("#", new AddressSettings().setAutoCreateAddresses(true));
+    }
+} catch (Throwable e) {
+    // need to ignore this for 1.4
+    e.printStackTrace();
+}
+
+jmsConfiguration = new JMSConfigurationImpl();
+
+server = new EmbeddedJMS();
+server.setConfiguration(configuration);
+server.setJmsConfiguration(jmsConfiguration);
+server.start();
+
+server.getJMSServerManager().createTopic(true, "topic");
+server.getJMSServerManager().createQueue(true, "queue", null, true);

http://git-wip-us.apache.org/repos/asf/activemq-artemis/blob/9c8bf2f2/tests/compatibility-tests/src/main/resources/prefixSendAckTest/artemisServer.groovy
----------------------------------------------------------------------
diff --git a/tests/compatibility-tests/src/main/resources/prefixSendAckTest/artemisServer.groovy b/tests/compatibility-tests/src/main/resources/prefixSendAckTest/artemisServer.groovy
new file mode 100644
index 0000000..7663c65
--- /dev/null
+++ b/tests/compatibility-tests/src/main/resources/prefixSendAckTest/artemisServer.groovy
@@ -0,0 +1,60 @@
+package servers
+/*
+ * 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.
+ */
+
+// starts an artemis server
+
+import org.apache.activemq.artemis.core.config.impl.ConfigurationImpl;
+import org.apache.activemq.artemis.core.server.JournalType;
+import org.apache.activemq.artemis.core.settings.impl.AddressSettings;
+import org.apache.activemq.artemis.jms.server.config.impl.JMSConfigurationImpl;
+import org.apache.activemq.artemis.jms.server.embedded.EmbeddedJMS
+
+
+String folder = arg[0];
+String id = arg[1];
+String type = arg[2];
+String producer = arg[3];
+String consumer = arg[4];
+
+println("type = " + type);
+
+configuration = new ConfigurationImpl();
+configuration.setJournalType(JournalType.NIO);
+System.out.println("folder:: " + folder);
+configuration.setBrokerInstance(new File(folder + "/" + id));
+configuration.addAcceptorConfiguration("artemis", "tcp://0.0.0.0:61616");
+configuration.setSecurityEnabled(false);
+configuration.setPersistenceEnabled(persistent);
+try {
+    if (!type.startsWith("ARTEMIS-1")) {
+        configuration.addAddressesSetting("#", new AddressSettings().setAutoCreateAddresses(true));
+    }
+} catch (Throwable e) {
+    // need to ignore this for 1.4
+    e.printStackTrace();
+}
+
+jmsConfiguration = new JMSConfigurationImpl();
+
+server = new EmbeddedJMS();
+server.setConfiguration(configuration);
+server.setJmsConfiguration(jmsConfiguration);
+server.start();
+
+server.getJMSServerManager().createTopic(true, "topic");
+server.getJMSServerManager().createQueue(true, "jms.queue.queue", null, true);

http://git-wip-us.apache.org/repos/asf/activemq-artemis/blob/9c8bf2f2/tests/compatibility-tests/src/main/resources/prefixSendAckTest/sendAckMessages.groovy
----------------------------------------------------------------------
diff --git a/tests/compatibility-tests/src/main/resources/prefixSendAckTest/sendAckMessages.groovy b/tests/compatibility-tests/src/main/resources/prefixSendAckTest/sendAckMessages.groovy
new file mode 100644
index 0000000..9c945a6
--- /dev/null
+++ b/tests/compatibility-tests/src/main/resources/prefixSendAckTest/sendAckMessages.groovy
@@ -0,0 +1,105 @@
+package meshTest
+
+import org.apache.activemq.artemis.tests.compatibility.GroovyRun
+
+import javax.jms.*
+import java.util.concurrent.CountDownLatch
+import java.util.concurrent.TimeUnit
+
+/*
+ * 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.
+ */
+
+// starts an artemis server
+String serverType = arg[0];
+String clientType = arg[1];
+String operation = arg[2];
+
+
+String queueName
+if (clientType.equals(GroovyRun.SNAPSHOT) || clientType.equals(GroovyRun.TWO_FOUR)) {
+    queueName = "jms.queue.queue";
+} else {
+    queueName = "queue";
+}
+
+
+String textBody = "a rapadura e doce mas nao e mole nao";
+
+println("serverType " + serverType);
+
+if (clientType.startsWith("ARTEMIS")) {
+    // Can't depend directly on artemis, otherwise it wouldn't compile in hornetq
+    GroovyRun.evaluate("clients/artemisClient.groovy", "serverArg", serverType);
+} else {
+    // Can't depend directly on hornetq, otherwise it wouldn't compile in artemis
+    GroovyRun.evaluate("clients/hornetqClient.groovy", "serverArg");
+}
+
+
+Connection connection = cf.createConnection();
+Session session = connection.createSession(false, Session.AUTO_ACKNOWLEDGE);
+Queue queue = session.createQueue(queueName);
+
+if (operation.equals("sendAckMessages")) {
+
+    CountDownLatch latch = new CountDownLatch(10);
+
+    CompletionListener completionListener = new CompletionListener() {
+        @Override
+        void onCompletion(Message message) {
+            latch.countDown();
+        }
+
+        @Override
+        void onException(Message message, Exception exception) {
+
+        }
+    }
+
+    MessageProducer producer = session.createProducer(queue);
+    producer.setDeliveryMode(DeliveryMode.PERSISTENT);
+    for (int i = 0; i < 10; i++) {
+        producer.send(session.createTextMessage(textBody + i), completionListener);
+    }
+
+    GroovyRun.assertTrue(latch.await(10, TimeUnit.SECONDS));
+
+    System.out.println("Sending messages");
+    connection.close();
+    System.out.println("Message sent");
+} else if (operation.equals("receiveMessages")) {
+    MessageConsumer consumer = session.createConsumer(queue);
+    connection.start();
+
+    System.out.println("Receiving messages");
+
+    for (int i = 0; i < 10; i++) {
+        TextMessage message = consumer.receive(1000);
+        GroovyRun.assertNotNull(message);
+        GroovyRun.assertEquals(textBody + i, message.getText());
+    }
+
+    GroovyRun.assertNull(consumer.receiveNoWait());
+    connection.close();
+    System.out.println("Message received");
+} else {
+    throw new RuntimeException("Invalid operation " + operation);
+}
+
+
+
+

http://git-wip-us.apache.org/repos/asf/activemq-artemis/blob/9c8bf2f2/tests/compatibility-tests/src/main/resources/servers/artemisServer.groovy
----------------------------------------------------------------------
diff --git a/tests/compatibility-tests/src/main/resources/servers/artemisServer.groovy b/tests/compatibility-tests/src/main/resources/servers/artemisServer.groovy
index 9c15a08..fe58505 100644
--- a/tests/compatibility-tests/src/main/resources/servers/artemisServer.groovy
+++ b/tests/compatibility-tests/src/main/resources/servers/artemisServer.groovy
@@ -38,7 +38,7 @@ configuration = new ConfigurationImpl();
 configuration.setJournalType(JournalType.NIO);
 System.out.println("folder:: " + folder);
 configuration.setBrokerInstance(new File(folder + "/" + id));
-configuration.addAcceptorConfiguration("artemis", "tcp://0.0.0.0:61616");
+configuration.addAcceptorConfiguration("artemis", "tcp://0.0.0.0:61616?anycastPrefix=jms.queue.&multicastPrefix=jms.topic.");
 configuration.setSecurityEnabled(false);
 configuration.setPersistenceEnabled(persistent);
 try {

http://git-wip-us.apache.org/repos/asf/activemq-artemis/blob/9c8bf2f2/tests/compatibility-tests/src/test/java/org/apache/activemq/artemis/tests/compatibility/ExportImportTest.java
----------------------------------------------------------------------
diff --git a/tests/compatibility-tests/src/test/java/org/apache/activemq/artemis/tests/compatibility/ExportImportTest.java b/tests/compatibility-tests/src/test/java/org/apache/activemq/artemis/tests/compatibility/ExportImportTest.java
index 69fac32..e70d02c 100644
--- a/tests/compatibility-tests/src/test/java/org/apache/activemq/artemis/tests/compatibility/ExportImportTest.java
+++ b/tests/compatibility-tests/src/test/java/org/apache/activemq/artemis/tests/compatibility/ExportImportTest.java
@@ -46,6 +46,7 @@ import static org.apache.activemq.artemis.tests.compatibility.GroovyRun.SNAPSHOT
  */
 @RunWith(Parameterized.class)
 public class ExportImportTest extends VersionedBaseTest {
+   private String serverScriptToUse;
 
    // this will ensure that all tests in this class are run twice,
    // once with "true" passed to the class' constructor and once with "false"
@@ -107,6 +108,9 @@ public class ExportImportTest extends VersionedBaseTest {
    public void internalSendReceive(boolean legacyPrefixes) throws Throwable {
       setVariable(senderClassloader, "legacy", false);
       setVariable(senderClassloader, "persistent", true);
+      if (legacyPrefixes) {
+         serverScriptToUse = "exportimport/artemisServer.groovy";
+      }
       startServer(serverFolder.getRoot(), senderClassloader, "sender");
       evaluate(senderClassloader, "meshTest/sendMessages.groovy", server, sender, "sendAckMessages");
       stopServer(senderClassloader);
@@ -133,5 +137,9 @@ public class ExportImportTest extends VersionedBaseTest {
       }
    }
 
+   @Override
+   public String getServerScriptToUse() {
+      return serverScriptToUse;
+   }
 }
 

http://git-wip-us.apache.org/repos/asf/activemq-artemis/blob/9c8bf2f2/tests/compatibility-tests/src/test/java/org/apache/activemq/artemis/tests/compatibility/PrefixSendAckTest.java
----------------------------------------------------------------------
diff --git a/tests/compatibility-tests/src/test/java/org/apache/activemq/artemis/tests/compatibility/PrefixSendAckTest.java b/tests/compatibility-tests/src/test/java/org/apache/activemq/artemis/tests/compatibility/PrefixSendAckTest.java
new file mode 100644
index 0000000..3bdb23a
--- /dev/null
+++ b/tests/compatibility-tests/src/test/java/org/apache/activemq/artemis/tests/compatibility/PrefixSendAckTest.java
@@ -0,0 +1,59 @@
+/*
+ * 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.compatibility;
+
+import java.util.ArrayList;
+import java.util.Collection;
+import java.util.List;
+
+import org.junit.Test;
+import org.junit.runner.RunWith;
+import org.junit.runners.Parameterized;
+
+import static org.apache.activemq.artemis.tests.compatibility.GroovyRun.HORNETQ_247;
+import static org.apache.activemq.artemis.tests.compatibility.GroovyRun.ONE_FIVE;
+import static org.apache.activemq.artemis.tests.compatibility.GroovyRun.SNAPSHOT;
+import static org.apache.activemq.artemis.tests.compatibility.GroovyRun.TWO_FOUR;
+
+@RunWith(Parameterized.class)
+public class PrefixSendAckTest extends ServerBaseTest {
+   @Parameterized.Parameters(name = "server={0}, producer={1}, consumer={2}")
+   public static Collection getParameters() {
+      List<Object[]> combinations = new ArrayList<>();
+
+      combinations.addAll(combinatory(new Object[]{SNAPSHOT}, new Object[]{ONE_FIVE, HORNETQ_247, TWO_FOUR, SNAPSHOT}, new Object[]{ONE_FIVE, HORNETQ_247, TWO_FOUR, SNAPSHOT}));
+      return combinations;
+   }
+
+   public PrefixSendAckTest(String server, String sender, String receiver) throws Exception {
+      super(server, sender, receiver);
+   }
+
+   @Test
+   public void testSendReceive() throws Throwable {
+      evaluate(senderClassloader,  "prefixSendAckTest/sendAckMessages.groovy", server, sender, "sendAckMessages");
+      evaluate(receiverClassloader,  "prefixSendAckTest/sendAckMessages.groovy", server, receiver, "receiveMessages");
+   }
+
+   @Override
+   public String getServerScriptToUse() {
+      return "prefixSendAckTest/artemisServer.groovy";
+   }
+
+}
+

http://git-wip-us.apache.org/repos/asf/activemq-artemis/blob/9c8bf2f2/tests/compatibility-tests/src/test/java/org/apache/activemq/artemis/tests/compatibility/VersionedBaseTest.java
----------------------------------------------------------------------
diff --git a/tests/compatibility-tests/src/test/java/org/apache/activemq/artemis/tests/compatibility/VersionedBaseTest.java b/tests/compatibility-tests/src/test/java/org/apache/activemq/artemis/tests/compatibility/VersionedBaseTest.java
index 8dc3302..e2b9648 100644
--- a/tests/compatibility-tests/src/test/java/org/apache/activemq/artemis/tests/compatibility/VersionedBaseTest.java
+++ b/tests/compatibility-tests/src/test/java/org/apache/activemq/artemis/tests/compatibility/VersionedBaseTest.java
@@ -194,7 +194,9 @@ public abstract class VersionedBaseTest {
       System.out.println("Folder::" + folder);
 
       String scriptToUse;
-      if (server.startsWith("ARTEMIS")) {
+      if (getServerScriptToUse() != null && getServerScriptToUse().length() != 0) {
+         scriptToUse = getServerScriptToUse();
+      } else if (server.startsWith("ARTEMIS")) {
          scriptToUse = "servers/artemisServer.groovy";
       } else {
          scriptToUse = "servers/hornetqServer.groovy";
@@ -206,4 +208,8 @@ public abstract class VersionedBaseTest {
    public void stopServer(ClassLoader loader) throws Throwable {
       execute(loader, "server.stop()");
    }
+
+   public String getServerScriptToUse() {
+      return null;
+   }
 }


[2/2] activemq-artemis git commit: This closes #1830

Posted by cl...@apache.org.
This closes #1830


Project: http://git-wip-us.apache.org/repos/asf/activemq-artemis/repo
Commit: http://git-wip-us.apache.org/repos/asf/activemq-artemis/commit/5c6f0c6d
Tree: http://git-wip-us.apache.org/repos/asf/activemq-artemis/tree/5c6f0c6d
Diff: http://git-wip-us.apache.org/repos/asf/activemq-artemis/diff/5c6f0c6d

Branch: refs/heads/master
Commit: 5c6f0c6df32c130201470cc1a9473b7a6a54475b
Parents: 3d79a08 9c8bf2f
Author: Clebert Suconic <cl...@apache.org>
Authored: Wed Jan 31 18:13:30 2018 -0500
Committer: Clebert Suconic <cl...@apache.org>
Committed: Wed Jan 31 18:13:30 2018 -0500

----------------------------------------------------------------------
 .../artemis/cli/commands/etc/broker.xml         |   4 +
 .../cli/commands/etc/hornetq-acceptor.txt       |   2 +-
 .../impl/wireformat/QueueAbstractPacket.java    |  81 ++++++--------
 .../artemis/api/core/QueueAbstractTest.java     |  48 ---------
 .../hornetq/HornetQProtocolManager.java         |   5 +
 .../hornetq/HornetQProtocolManagerFactory.java  |   5 +-
 .../core/ServerSessionPacketHandler.java        |   8 +-
 .../core/impl/ActiveMQPacketHandler.java        |   9 +-
 .../protocol/core/impl/CoreProtocolManager.java |   2 +-
 .../addressConfig/artemisServer.groovy          |   2 +-
 .../resources/exportimport/artemisServer.groovy |  60 +++++++++++
 .../prefixSendAckTest/artemisServer.groovy      |  60 +++++++++++
 .../prefixSendAckTest/sendAckMessages.groovy    | 105 +++++++++++++++++++
 .../main/resources/servers/artemisServer.groovy |   2 +-
 .../tests/compatibility/ExportImportTest.java   |   8 ++
 .../tests/compatibility/PrefixSendAckTest.java  |  59 +++++++++++
 .../tests/compatibility/VersionedBaseTest.java  |   8 +-
 17 files changed, 351 insertions(+), 117 deletions(-)
----------------------------------------------------------------------