You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@activemq.apache.org by ta...@apache.org on 2016/04/06 17:23:42 UTC

[1/2] activemq git commit: https://issues.apache.org/jira/browse/AMQ-6215 - support 0 maxBrowsePageSize and maxExpirePageSize such that lazyDispatch ensures highest priority messages is available to a pull consumer

Repository: activemq
Updated Branches:
  refs/heads/activemq-5.13.x 8393e6b8e -> 6c7929854


https://issues.apache.org/jira/browse/AMQ-6215 - support 0 maxBrowsePageSize and maxExpirePageSize such that lazyDispatch ensures highest priority messages is available to a pull consumer

(cherry picked from commit a3a8c1c5256aa8ea1067afe3e1586832e5aa1821)


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

Branch: refs/heads/activemq-5.13.x
Commit: 06972183f9dc8ed575e070d553f189552b027601
Parents: 8393e6b
Author: gtully <ga...@gmail.com>
Authored: Tue Apr 5 13:22:23 2016 +0100
Committer: Timothy Bish <ta...@gmail.com>
Committed: Wed Apr 6 11:20:07 2016 -0400

----------------------------------------------------------------------
 .../activemq/broker/region/BaseDestination.java |   2 +-
 .../apache/activemq/broker/region/Queue.java    |  83 ++--
 ...eueZeroPrefetchLazyDispatchPriorityTest.java | 393 +++++++++++++++++++
 3 files changed, 439 insertions(+), 39 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/activemq/blob/06972183/activemq-broker/src/main/java/org/apache/activemq/broker/region/BaseDestination.java
----------------------------------------------------------------------
diff --git a/activemq-broker/src/main/java/org/apache/activemq/broker/region/BaseDestination.java b/activemq-broker/src/main/java/org/apache/activemq/broker/region/BaseDestination.java
index 5ae2d28..75f2ee0 100755
--- a/activemq-broker/src/main/java/org/apache/activemq/broker/region/BaseDestination.java
+++ b/activemq-broker/src/main/java/org/apache/activemq/broker/region/BaseDestination.java
@@ -316,7 +316,7 @@ public abstract class BaseDestination implements Destination {
 
     @Override
     public int getMaxBrowsePageSize() {
-        return this.maxBrowsePageSize > 0 ? this.maxBrowsePageSize : getMaxPageSize();
+        return this.maxBrowsePageSize;
     }
 
     @Override

http://git-wip-us.apache.org/repos/asf/activemq/blob/06972183/activemq-broker/src/main/java/org/apache/activemq/broker/region/Queue.java
----------------------------------------------------------------------
diff --git a/activemq-broker/src/main/java/org/apache/activemq/broker/region/Queue.java b/activemq-broker/src/main/java/org/apache/activemq/broker/region/Queue.java
index cf53b51..786640e 100755
--- a/activemq-broker/src/main/java/org/apache/activemq/broker/region/Queue.java
+++ b/activemq-broker/src/main/java/org/apache/activemq/broker/region/Queue.java
@@ -1144,17 +1144,17 @@ public class Queue extends BaseDestination implements Task, UsageListener, Index
         final ConnectionContext connectionContext = createConnectionContext();
         try {
             int maxPageInAttempts = 1;
-            messagesLock.readLock().lock();
-            try {
-                maxPageInAttempts += (messages.size() / getMaxPageSize());
-            } finally {
-                messagesLock.readLock().unlock();
+            if (max > 0) {
+                messagesLock.readLock().lock();
+                try {
+                    maxPageInAttempts += (messages.size() / max);
+                } finally {
+                    messagesLock.readLock().unlock();
+                }
+                while (shouldPageInMoreForBrowse(max) && maxPageInAttempts-- > 0) {
+                    pageInMessages(!memoryUsage.isFull(110), max);
+                }
             }
-
-            while (shouldPageInMoreForBrowse(max) && maxPageInAttempts-- > 0) {
-                pageInMessages(!memoryUsage.isFull(110));
-            };
-
             doBrowseList(browseList, max, dispatchPendingList, pagedInPendingDispatchLock, connectionContext, "redeliveredWaitingDispatch+pagedInPendingDispatch");
             doBrowseList(browseList, max, pagedInMessages, pagedInMessagesLock, connectionContext, "pagedInMessages");
 
@@ -1262,7 +1262,7 @@ public class Queue extends BaseDestination implements Task, UsageListener, Index
         List<MessageReference> list = null;
         long originalMessageCount = this.destinationStatistics.getMessages().getCount();
         do {
-            doPageIn(true, false);  // signal no expiry processing needed.
+            doPageIn(true, false, getMaxPageSize());  // signal no expiry processing needed.
             pagedInMessagesLock.readLock().lock();
             try {
                 list = new ArrayList<MessageReference>(pagedInMessages.values());
@@ -1630,7 +1630,7 @@ public class Queue extends BaseDestination implements Task, UsageListener, Index
 
             if (pageInMoreMessages || hasBrowsers || !dispatchPendingList.hasRedeliveries()) {
                 try {
-                    pageInMessages(hasBrowsers);
+                    pageInMessages(hasBrowsers && getMaxBrowsePageSize() > 0, getMaxPageSize());
                 } catch (Throwable e) {
                     LOG.error("Failed to page in more queue messages ", e);
                 }
@@ -1895,11 +1895,11 @@ public class Queue extends BaseDestination implements Task, UsageListener, Index
     }
 
     private void doPageIn(boolean force) throws Exception {
-        doPageIn(force, true);
+        doPageIn(force, true, getMaxPageSize());
     }
 
-    private void doPageIn(boolean force, boolean processExpired) throws Exception {
-        PendingList newlyPaged = doPageInForDispatch(force, processExpired);
+    private void doPageIn(boolean force, boolean processExpired, int maxPageSize) throws Exception {
+        PendingList newlyPaged = doPageInForDispatch(force, processExpired, maxPageSize);
         pagedInPendingDispatchLock.writeLock().lock();
         try {
             if (dispatchPendingList.isEmpty()) {
@@ -1917,11 +1917,11 @@ public class Queue extends BaseDestination implements Task, UsageListener, Index
         }
     }
 
-    private PendingList doPageInForDispatch(boolean force, boolean processExpired) throws Exception {
+    private PendingList doPageInForDispatch(boolean force, boolean processExpired, int maxPageSize) throws Exception {
         List<QueueMessageReference> result = null;
         PendingList resultList = null;
 
-        int toPageIn = Math.min(getMaxPageSize(), messages.size());
+        int toPageIn = Math.min(maxPageSize, messages.size());
         int pagedInPendingSize = 0;
         pagedInPendingDispatchLock.readLock().lock();
         try {
@@ -1929,24 +1929,29 @@ public class Queue extends BaseDestination implements Task, UsageListener, Index
         } finally {
             pagedInPendingDispatchLock.readLock().unlock();
         }
-
-        LOG.debug("{} toPageIn: {}, Inflight: {}, pagedInMessages.size {}, pagedInPendingDispatch.size {}, enqueueCount: {}, dequeueCount: {}, memUsage:{}",
-                new Object[]{
-                        this,
-                        toPageIn,
-                        destinationStatistics.getInflight().getCount(),
-                        pagedInMessages.size(),
-                        pagedInPendingSize,
-                        destinationStatistics.getEnqueues().getCount(),
-                        destinationStatistics.getDequeues().getCount(),
-                        getMemoryUsage().getUsage()
-                });
         if (isLazyDispatch() && !force) {
             // Only page in the minimum number of messages which can be
             // dispatched immediately.
             toPageIn = Math.min(getConsumerMessageCountBeforeFull(), toPageIn);
         }
-        if (toPageIn > 0 && (force || (!consumers.isEmpty() && pagedInPendingSize < getMaxPageSize()))) {
+
+        if (LOG.isDebugEnabled()) {
+            LOG.debug("{} toPageIn: {}, force:{}, Inflight: {}, pagedInMessages.size {}, pagedInPendingDispatch.size {}, enqueueCount: {}, dequeueCount: {}, memUsage:{}, maxPageSize:{}",
+                    new Object[]{
+                            this,
+                            toPageIn,
+                            force,
+                            destinationStatistics.getInflight().getCount(),
+                            pagedInMessages.size(),
+                            pagedInPendingSize,
+                            destinationStatistics.getEnqueues().getCount(),
+                            destinationStatistics.getDequeues().getCount(),
+                            getMemoryUsage().getUsage(),
+                            maxPageSize
+                    });
+        }
+
+        if (toPageIn > 0 && (force || (haveRealConsumer() && pagedInPendingSize < maxPageSize))) {
             int count = 0;
             result = new ArrayList<QueueMessageReference>(toPageIn);
             messagesLock.writeLock().lock();
@@ -1954,7 +1959,7 @@ public class Queue extends BaseDestination implements Task, UsageListener, Index
                 try {
                     messages.setMaxBatchSize(toPageIn);
                     messages.reset();
-                    while (messages.hasNext() && count < toPageIn) {
+                    while (count < toPageIn && messages.hasNext()) {
                         MessageReference node = messages.next();
                         messages.remove();
 
@@ -2013,6 +2018,10 @@ public class Queue extends BaseDestination implements Task, UsageListener, Index
         return resultList;
     }
 
+    private final boolean haveRealConsumer() {
+        return consumers.size() - browserDispatches.size() > 0;
+    }
+
     private void doDispatch(PendingList list) throws Exception {
         boolean doWakeUp = false;
 
@@ -2173,8 +2182,8 @@ public class Queue extends BaseDestination implements Task, UsageListener, Index
         subs.getConsumerInfo().incrementAssignedGroupCount(destination);
     }
 
-    protected void pageInMessages(boolean force) throws Exception {
-        doDispatch(doPageInForDispatch(force, true));
+    protected void pageInMessages(boolean force, int maxPageSize) throws Exception {
+        doDispatch(doPageInForDispatch(force, true, maxPageSize));
     }
 
     private void addToConsumerList(Subscription sub) {
@@ -2192,20 +2201,18 @@ public class Queue extends BaseDestination implements Task, UsageListener, Index
 
     private int getConsumerMessageCountBeforeFull() throws Exception {
         int total = 0;
-        boolean zeroPrefetch = false;
         consumersLock.readLock().lock();
         try {
             for (Subscription s : consumers) {
-                zeroPrefetch |= s.getPrefetchSize() == 0;
+                if (s.isBrowser()) {
+                    continue;
+                }
                 int countBeforeFull = s.countBeforeFull();
                 total += countBeforeFull;
             }
         } finally {
             consumersLock.readLock().unlock();
         }
-        if (total == 0 && zeroPrefetch) {
-            total = 1;
-        }
         return total;
     }
 

http://git-wip-us.apache.org/repos/asf/activemq/blob/06972183/activemq-unit-tests/src/test/java/org/apache/activemq/usecases/QueueZeroPrefetchLazyDispatchPriorityTest.java
----------------------------------------------------------------------
diff --git a/activemq-unit-tests/src/test/java/org/apache/activemq/usecases/QueueZeroPrefetchLazyDispatchPriorityTest.java b/activemq-unit-tests/src/test/java/org/apache/activemq/usecases/QueueZeroPrefetchLazyDispatchPriorityTest.java
new file mode 100644
index 0000000..cff3bee
--- /dev/null
+++ b/activemq-unit-tests/src/test/java/org/apache/activemq/usecases/QueueZeroPrefetchLazyDispatchPriorityTest.java
@@ -0,0 +1,393 @@
+/**
+ * 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
+ * <p/>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p/>
+ * 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.usecases;
+
+import java.util.ArrayList;
+import java.util.Enumeration;
+import java.util.List;
+import javax.jms.BytesMessage;
+import javax.jms.Connection;
+import javax.jms.ConnectionFactory;
+import javax.jms.Message;
+import javax.jms.MessageConsumer;
+import javax.jms.MessageProducer;
+import javax.jms.QueueBrowser;
+import javax.jms.Session;
+import junit.framework.TestCase;
+import org.apache.activemq.ActiveMQConnectionFactory;
+import org.apache.activemq.broker.BrokerService;
+import org.apache.activemq.broker.region.policy.PolicyEntry;
+import org.apache.activemq.broker.region.policy.PolicyMap;
+import org.apache.activemq.command.ActiveMQQueue;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+
+public class QueueZeroPrefetchLazyDispatchPriorityTest extends TestCase {
+
+    static final Logger LOG = LoggerFactory.getLogger(QueueZeroPrefetchLazyDispatchPriorityTest.class);
+    private BrokerService broker;
+    public static final byte[] PAYLOAD = new byte[]{0, 1, 2, 3, 4, 5, 6, 7, 8, 9};
+
+
+    protected void setUp() throws Exception {
+        broker = createBroker();
+        broker.start();
+        broker.waitUntilStarted();
+    }
+
+    protected void tearDown() throws Exception {
+
+        if (broker != null) {
+            broker.stop();
+        }
+    }
+
+
+    public void testPriorityMessages() throws Exception {
+
+
+        for (int i = 0; i < 5; i++) {
+
+
+            //send 4 message priority MEDIUM
+            produceMessages(4, 4, "TestQ");
+
+
+            //send 1 message priority HIGH
+            produceMessages(1, 5, "TestQ");
+
+
+            LOG.info("On iteration " + i);
+
+
+            Thread.sleep(500);
+
+
+            // consume messages
+            ArrayList<Message> consumeList = consumeMessages("TestQ");
+            LOG.info("Consumed list " + consumeList.size());
+
+
+            // compare lists
+            assertEquals("message 1 should be priority high", 5, consumeList.get(0).getJMSPriority());
+            assertEquals("message 2 should be priority medium", 4, consumeList.get(1).getJMSPriority());
+            assertEquals("message 3 should be priority medium", 4, consumeList.get(2).getJMSPriority());
+            assertEquals("message 4 should be priority medium", 4, consumeList.get(3).getJMSPriority());
+            assertEquals("message 5 should be priority medium", 4, consumeList.get(4).getJMSPriority());
+        }
+
+    }
+
+
+    public void testPriorityMessagesMoreThanPageSize() throws Exception {
+
+
+        final int numToSend = 450;
+        for (int i = 0; i < 5; i++) {
+
+            produceMessages(numToSend - 1, 4, "TestQ");
+
+            // ensure we get expiry processing
+            Thread.sleep(700);
+
+
+            //send 1 message priority HIGH
+            produceMessages(1, 5, "TestQ");
+
+            Thread.sleep(500);
+
+            LOG.info("On iteration " + i);
+
+            // consume messages
+            ArrayList<Message> consumeList = consumeMessages("TestQ");
+            LOG.info("Consumed list " + consumeList.size());
+
+
+            // compare lists
+            assertEquals("message 1 should be priority high", 5, consumeList.get(0).getJMSPriority());
+            for (int j = 1; j < (numToSend - 1); j++) {
+                assertEquals("message " + j + " should be priority medium", 4, consumeList.get(j).getJMSPriority());
+            }
+        }
+
+    }
+
+
+    public void testLongLivedPriorityConsumer() throws Exception {
+
+        final int numToSend = 150;
+
+        ConnectionFactory connectionFactory = new ActiveMQConnectionFactory(broker.getTransportConnectorByScheme("tcp").getPublishableConnectString());
+        Connection connection = connectionFactory.createConnection();
+        try {
+            Session session = connection.createSession(false, Session.AUTO_ACKNOWLEDGE);
+            MessageConsumer consumer = session.createConsumer(new ActiveMQQueue("TestQ"));
+            connection.start();
+
+            for (int i = 0; i < 5; i++) {
+
+                produceMessages(numToSend - 1, 4, "TestQ");
+
+                //send 1 message priority HIGH
+                produceMessages(1, 5, "TestQ");
+
+                Message message = consumer.receive(4000);
+
+                assertEquals("message should be priority high", 5, message.getJMSPriority());
+
+            }
+        } finally {
+            connection.close();
+        }
+
+        ArrayList<Message> consumeList = consumeMessages("TestQ");
+        LOG.info("Consumed list " + consumeList.size());
+
+        for (Message message : consumeList) {
+            assertEquals("should be priority medium", 4, message.getJMSPriority());
+        }
+
+    }
+
+
+    public void testPriorityMessagesWithJmsBrowser() throws Exception {
+
+
+        final int numToSend = 250;
+        for (int i = 0; i < 5; i++) {
+
+            produceMessages(numToSend - 1, 4, "TestQ");
+
+            ArrayList<Message> browsed = browseMessages("TestQ");
+
+            LOG.info("Browsed: " + browsed.size());
+
+            //send 1 message priority HIGH
+            produceMessages(1, 5, "TestQ");
+
+            Thread.sleep(500);
+
+            LOG.info("On iteration " + i);
+
+            Message message = consumeOneMessage("TestQ");
+            assertNotNull(message);
+            assertEquals(5, message.getJMSPriority());
+
+            // consume messages
+            ArrayList<Message> consumeList = consumeMessages("TestQ");
+            LOG.info("Consumed list " + consumeList.size());
+
+
+            // compare lists
+            //assertEquals("Iteration: " + i +", message 1 should be priority high", 5, consumeList.get(0).getJMSPriority());
+            for (int j = 1; j < (numToSend - 1); j++) {
+                assertEquals("Iteration: " + i + ", message " + j + " should be priority medium", 4, consumeList.get(j).getJMSPriority());
+            }
+        }
+
+    }
+
+    public void testJmsBrowserGetsPagedIn() throws Exception {
+
+
+        final int numToSend = 10;
+        for (int i = 0; i < 10; i++) {
+
+            produceMessages(numToSend, 4, "TestQ");
+
+            ArrayList<Message> browsed = browseMessages("TestQ");
+
+            LOG.info("Browsed: " + browsed.size());
+
+            assertEquals(0, browsed.size());
+
+            Message message = consumeOneMessage("TestQ", Session.CLIENT_ACKNOWLEDGE);
+            assertNotNull(message);
+
+            browsed = browseMessages("TestQ");
+
+            LOG.info("Browsed: " + browsed.size());
+
+            assertEquals("see only the paged in for pull", 1, browsed.size());
+
+            // consume messages
+            ArrayList<Message> consumeList = consumeMessages("TestQ");
+            LOG.info("Consumed list " + consumeList.size());
+            assertEquals(numToSend, consumeList.size());
+
+        }
+
+    }
+
+
+    private void produceMessages(int numberOfMessages, int priority, String queueName) throws Exception {
+
+        ActiveMQConnectionFactory connectionFactory = new ActiveMQConnectionFactory(broker.getTransportConnectorByScheme("tcp").getPublishableConnectString());
+        connectionFactory.setConnectionIDPrefix("pri-" + priority);
+        Connection connection = connectionFactory.createConnection();
+        try {
+            Session session = connection.createSession(false, Session.AUTO_ACKNOWLEDGE);
+            MessageProducer producer = session.createProducer(new ActiveMQQueue(queueName));
+            connection.start();
+
+
+            for (int i = 0; i < numberOfMessages; i++) {
+                BytesMessage m = session.createBytesMessage();
+                m.writeBytes(PAYLOAD);
+                m.setJMSPriority(priority);
+                producer.send(m, Message.DEFAULT_DELIVERY_MODE, m.getJMSPriority(), Message.DEFAULT_TIME_TO_LIVE);
+            }
+
+        } finally {
+
+            if (connection != null) {
+                connection.close();
+            }
+        }
+    }
+
+
+    private ArrayList<Message> consumeMessages(String queueName) throws Exception {
+
+        ArrayList<Message> returnedMessages = new ArrayList<Message>();
+
+        ConnectionFactory connectionFactory = new ActiveMQConnectionFactory(broker.getTransportConnectorByScheme("tcp").getPublishableConnectString());
+        Connection connection = connectionFactory.createConnection();
+        try {
+            Session session = connection.createSession(false, Session.AUTO_ACKNOWLEDGE);
+            MessageConsumer consumer = session.createConsumer(new ActiveMQQueue(queueName));
+            connection.start();
+            boolean finished = false;
+
+            while (!finished) {
+
+                Message message = consumer.receive(1000);
+                if (message == null) {
+                    finished = true;
+                }
+
+                if (message != null) {
+                    returnedMessages.add(message);
+                }
+
+            }
+
+            consumer.close();
+            return returnedMessages;
+
+        } finally {
+
+            if (connection != null) {
+                connection.close();
+            }
+        }
+
+
+    }
+
+    private Message consumeOneMessage(String queueName) throws Exception {
+        return consumeOneMessage(queueName, Session.AUTO_ACKNOWLEDGE);
+    }
+
+    private Message consumeOneMessage(String queueName, int ackMode) throws Exception {
+
+        ConnectionFactory connectionFactory = new ActiveMQConnectionFactory(broker.getTransportConnectorByScheme("tcp").getPublishableConnectString());
+        Connection connection = connectionFactory.createConnection();
+        try {
+            Session session = connection.createSession(false, ackMode);
+            MessageConsumer consumer = session.createConsumer(new ActiveMQQueue(queueName));
+            connection.start();
+
+            return consumer.receive(1000);
+
+        } finally {
+
+            if (connection != null) {
+                connection.close();
+            }
+        }
+
+    }
+
+    private ArrayList<Message> browseMessages(String queueName) throws Exception {
+
+        ArrayList<Message> returnedMessages = new ArrayList<Message>();
+
+        ConnectionFactory connectionFactory = new ActiveMQConnectionFactory(broker.getTransportConnectorByScheme("tcp").getPublishableConnectString());
+        Connection connection = connectionFactory.createConnection();
+        try {
+            Session session = connection.createSession(false, Session.AUTO_ACKNOWLEDGE);
+            QueueBrowser consumer = session.createBrowser(new ActiveMQQueue(queueName));
+            connection.start();
+
+            Enumeration enumeration = consumer.getEnumeration();
+            while (enumeration.hasMoreElements()) {
+
+                Message message = (Message) enumeration.nextElement();
+                returnedMessages.add(message);
+
+            }
+
+            return returnedMessages;
+
+        } finally {
+
+            if (connection != null) {
+                connection.close();
+            }
+        }
+
+
+    }
+
+    private BrokerService createBroker() throws Exception {
+        BrokerService broker = new BrokerService();
+        broker.setDeleteAllMessagesOnStartup(true);
+
+        //add the policy entries
+        PolicyMap policyMap = new PolicyMap();
+        List<PolicyEntry> entries = new ArrayList<PolicyEntry>();
+        PolicyEntry pe = new PolicyEntry();
+
+        pe.setPrioritizedMessages(true);
+
+        pe.setExpireMessagesPeriod(500);
+
+        pe.setMaxPageSize(100);
+        pe.setMaxExpirePageSize(0);
+        pe.setMaxBrowsePageSize(0);
+
+        pe.setQueuePrefetch(0);
+        pe.setLazyDispatch(true);
+
+        pe.setOptimizedDispatch(true);
+
+        pe.setUseCache(false);
+
+        pe.setQueue(">");
+        entries.add(pe);
+        policyMap.setPolicyEntries(entries);
+        broker.setDestinationPolicy(policyMap);
+
+
+        broker.addConnector("tcp://0.0.0.0:0");
+        return broker;
+    }
+
+
+}
\ No newline at end of file


[2/2] activemq git commit: https://issues.apache.org/jira/browse/AMQ-6151 - retain list for redeliveries and combine for dispatch/iteration such that redeliveries retain per priority order after prefetch

Posted by ta...@apache.org.
https://issues.apache.org/jira/browse/AMQ-6151 - retain list for redeliveries and combine for dispatch/iteration such that redeliveries retain per priority order after prefetch

(cherry picked from commit 2a8218a9a8cfa74e1049249481f601e042f33358)


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

Branch: refs/heads/activemq-5.13.x
Commit: 6c79298541541526fedc6a0580c1b487d505212e
Parents: 0697218
Author: gtully <ga...@gmail.com>
Authored: Wed Apr 6 15:09:36 2016 +0100
Committer: Timothy Bish <ta...@gmail.com>
Committed: Wed Apr 6 11:20:21 2016 -0400

----------------------------------------------------------------------
 .../cursors/QueueDispatchPendingList.java       |  86 +++++---
 .../usecases/PriorityRedeliveryOrderTest.java   | 210 +++++++++++++++++++
 2 files changed, 267 insertions(+), 29 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/activemq/blob/6c792985/activemq-broker/src/main/java/org/apache/activemq/broker/region/cursors/QueueDispatchPendingList.java
----------------------------------------------------------------------
diff --git a/activemq-broker/src/main/java/org/apache/activemq/broker/region/cursors/QueueDispatchPendingList.java b/activemq-broker/src/main/java/org/apache/activemq/broker/region/cursors/QueueDispatchPendingList.java
index 385e2b8..788b5e5 100644
--- a/activemq-broker/src/main/java/org/apache/activemq/broker/region/cursors/QueueDispatchPendingList.java
+++ b/activemq-broker/src/main/java/org/apache/activemq/broker/region/cursors/QueueDispatchPendingList.java
@@ -40,7 +40,6 @@ public class QueueDispatchPendingList implements PendingList {
 
     private PendingList pagedInPendingDispatch = new OrderedPendingList();
     private PendingList redeliveredWaitingDispatch = new OrderedPendingList();
-    // when true use one PrioritizedPendingList for everything
     private boolean prioritized = false;
 
 
@@ -87,7 +86,7 @@ public class QueueDispatchPendingList implements PendingList {
     public PendingNode remove(MessageReference message) {
         if (pagedInPendingDispatch.contains(message)) {
             return pagedInPendingDispatch.remove(message);
-        }else if (redeliveredWaitingDispatch.contains(message)) {
+        } else if (redeliveredWaitingDispatch.contains(message)) {
             return redeliveredWaitingDispatch.remove(message);
         }
         return null;
@@ -105,31 +104,64 @@ public class QueueDispatchPendingList implements PendingList {
 
     @Override
     public Iterator<MessageReference> iterator() {
-        return new Iterator<MessageReference>() {
+        if (prioritized && hasRedeliveries()) {
+            final QueueDispatchPendingList delegate = this;
+            final PrioritizedPendingList  priorityOrderedRedeliveredAndPending = new PrioritizedPendingList();
+            priorityOrderedRedeliveredAndPending.addAll(redeliveredWaitingDispatch);
+            priorityOrderedRedeliveredAndPending.addAll(pagedInPendingDispatch);
 
-            Iterator<MessageReference> redeliveries = redeliveredWaitingDispatch.iterator();
-            Iterator<MessageReference> pendingDispatch = pagedInPendingDispatch.iterator();
-            Iterator<MessageReference> current = redeliveries;
+            return new Iterator<MessageReference>() {
 
+                Iterator<MessageReference> combinedIterator = priorityOrderedRedeliveredAndPending.iterator();
+                MessageReference current = null;
 
-            @Override
-            public boolean hasNext() {
-                if (!redeliveries.hasNext() && (current == redeliveries)) {
-                    current = pendingDispatch;
+                @Override
+                public boolean hasNext() {
+                    return combinedIterator.hasNext();
                 }
-                return current.hasNext();
-            }
-
-            @Override
-            public MessageReference next() {
-                return current.next();
-            }
-
-            @Override
-            public void remove() {
-                current.remove();
-            }
-        };
+
+                @Override
+                public MessageReference next() {
+                    current = combinedIterator.next();
+                    return current;
+                }
+
+                @Override
+                public void remove() {
+                    if (current!=null) {
+                        delegate.remove(current);
+                    }
+                }
+            };
+
+        } else {
+
+            return new Iterator<MessageReference>() {
+
+                Iterator<MessageReference> redeliveries = redeliveredWaitingDispatch.iterator();
+                Iterator<MessageReference> pendingDispatch = pagedInPendingDispatch.iterator();
+                Iterator<MessageReference> current = redeliveries;
+
+
+                @Override
+                public boolean hasNext() {
+                    if (!redeliveries.hasNext() && (current == redeliveries)) {
+                        current = pendingDispatch;
+                    }
+                    return current.hasNext();
+                }
+
+                @Override
+                public MessageReference next() {
+                    return current.next();
+                }
+
+                @Override
+                public void remove() {
+                    current.remove();
+                }
+            };
+        }
     }
 
     @Override
@@ -173,14 +205,10 @@ public class QueueDispatchPendingList implements PendingList {
     }
 
     public void addMessageForRedelivery(QueueMessageReference qmr) {
-        if (prioritized) {
-            pagedInPendingDispatch.addMessageLast(qmr);
-        } else {
-            redeliveredWaitingDispatch.addMessageLast(qmr);
-        }
+        redeliveredWaitingDispatch.addMessageLast(qmr);
     }
 
     public boolean hasRedeliveries(){
-        return prioritized ? !pagedInPendingDispatch.isEmpty() : !redeliveredWaitingDispatch.isEmpty();
+        return !redeliveredWaitingDispatch.isEmpty();
     }
 }

http://git-wip-us.apache.org/repos/asf/activemq/blob/6c792985/activemq-unit-tests/src/test/java/org/apache/activemq/usecases/PriorityRedeliveryOrderTest.java
----------------------------------------------------------------------
diff --git a/activemq-unit-tests/src/test/java/org/apache/activemq/usecases/PriorityRedeliveryOrderTest.java b/activemq-unit-tests/src/test/java/org/apache/activemq/usecases/PriorityRedeliveryOrderTest.java
new file mode 100644
index 0000000..35f68f5
--- /dev/null
+++ b/activemq-unit-tests/src/test/java/org/apache/activemq/usecases/PriorityRedeliveryOrderTest.java
@@ -0,0 +1,210 @@
+/**
+ * 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
+ * <p/>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p/>
+ * 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.usecases;
+
+import java.util.ArrayList;
+import java.util.List;
+import javax.jms.Connection;
+import javax.jms.ConnectionFactory;
+import javax.jms.Message;
+import javax.jms.MessageConsumer;
+import javax.jms.MessageProducer;
+import javax.jms.Session;
+import org.apache.activemq.ActiveMQConnection;
+import org.apache.activemq.ActiveMQConnectionFactory;
+import org.apache.activemq.broker.BrokerService;
+import org.apache.activemq.broker.region.policy.PolicyEntry;
+import org.apache.activemq.broker.region.policy.PolicyMap;
+import org.junit.After;
+import org.junit.Before;
+import org.junit.Test;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+
+import static org.junit.Assert.assertEquals;
+import static org.junit.Assert.fail;
+
+
+/**
+ * Sends X messages with a sequence number held in a JMS property "appId"
+ * Uses all priority 4 message (normal priority)
+ * closed the consumer connection multiple times so the already prefetched messages will be available
+ * for dispatch again.
+ */
+
+public class PriorityRedeliveryOrderTest {
+
+    private static final Logger LOG = LoggerFactory.getLogger(PriorityRedeliveryOrderTest.class);
+
+    private static final String DESTINATION = "testQ1";
+    private static final int MESSAGES_TO_SEND = 1000;
+    private static final int MESSAGES_PER_CONSUMER = 200;
+    private int consumedAppId = -1;
+    private int totalConsumed;
+    BrokerService broker;
+
+    @Before
+    public void createBroker() throws Exception {
+
+        broker = new BrokerService();
+        broker.setDeleteAllMessagesOnStartup(true);
+
+        PolicyMap policyMap = new PolicyMap();
+        List<PolicyEntry> entries = new ArrayList<PolicyEntry>();
+        PolicyEntry pe = new PolicyEntry();
+
+        pe.setPrioritizedMessages(true);
+
+        pe.setQueue(">");
+        entries.add(pe);
+        policyMap.setPolicyEntries(entries);
+        broker.setDestinationPolicy(policyMap);
+
+
+        broker.addConnector("tcp://0.0.0.0:0");
+
+        broker.start();
+        broker.waitUntilStarted();
+    }
+
+    @After
+    public void stopBroker() throws Exception {
+        broker.stop();
+        broker.waitUntilStopped();
+    }
+
+    @Test
+    public void testMessageDeliveryOrderAfterPrefetch() throws Exception {
+
+        //send X messages with with a sequence number number in the message property.
+        sendMessages(MESSAGES_TO_SEND);
+
+        for (int i = 0; i < (MESSAGES_TO_SEND / MESSAGES_PER_CONSUMER); i++) {
+            totalConsumed += consumeMessages(MESSAGES_PER_CONSUMER);
+        }
+        assertEquals("number of messages consumed should be equal to number of messages sent", MESSAGES_TO_SEND, totalConsumed);
+    }
+
+    private Long sendMessages(int messageCount) throws Exception {
+
+        long numberOfMessageSent = 0;
+
+        ConnectionFactory connectionFactory = new ActiveMQConnectionFactory(broker.getTransportConnectorByScheme("tcp").getPublishableConnectString());
+
+        Connection connection = connectionFactory.createConnection();
+        connection.start();
+
+        try {
+
+            Session producerSession = connection.createSession(true, Session.SESSION_TRANSACTED);
+            MessageProducer jmsProducer = producerSession.createProducer(producerSession.createQueue(DESTINATION));
+
+            Message sendMessage = producerSession.createTextMessage("test_message");
+
+            for (int i = 0; i < messageCount; i++) {
+
+                sendMessage.setIntProperty("appID", i);
+                jmsProducer.send(sendMessage);
+                producerSession.commit();
+                numberOfMessageSent++;
+
+            }
+
+            LOG.info(" Finished after producing : " + numberOfMessageSent);
+            return numberOfMessageSent;
+
+        } catch (Exception ex) {
+            LOG.info("Exception received producing ", ex);
+            LOG.info("finishing after exception :" + numberOfMessageSent);
+            return numberOfMessageSent;
+        } finally {
+            if (connection != null) {
+                connection.close();
+            }
+        }
+    }
+
+    /*
+     Ensure messages are consumed in the expected sequence
+     */
+
+    private int consumeMessages(int numberOfMessage) throws Exception {
+
+        LOG.info("Creating new consumer for:" + numberOfMessage);
+
+
+        int numberConsumedMessage = 0;
+        ConnectionFactory connectionFactory = new ActiveMQConnectionFactory(broker.getTransportConnectorByScheme("tcp").getPublishableConnectString());
+        ActiveMQConnection connection = (ActiveMQConnection) connectionFactory.createConnection();
+
+        try {
+
+            connection.start();
+            Session session = connection.createSession(true, Session.SESSION_TRANSACTED);
+            MessageConsumer jmsConsumer = session.createConsumer(session.createQueue(DESTINATION));
+            boolean consume = true;
+
+
+            while (consume) {
+
+                Message message = jmsConsumer.receive(4000);
+
+                if (message == null) {
+                    LOG.info("Break on:" + numberConsumedMessage);
+                    break;
+                }
+
+
+                int newAppId = message.getIntProperty("appID");
+
+                numberConsumedMessage++;
+
+                LOG.debug("Message newAppID" + newAppId);
+
+                //check it is next appID in sequence
+
+                if (newAppId != (consumedAppId + 1)) {
+                    fail(" newAppId is " + newAppId + " expected " + (consumedAppId + 1));
+                }
+
+                //increase next AppID
+                consumedAppId = newAppId;
+
+                session.commit();
+
+                if (numberConsumedMessage == numberOfMessage) {
+                    LOG.info("closing consumer after 200 message, consumedAppID is " + consumedAppId);
+                    return numberConsumedMessage;
+                }
+
+            }
+        } finally {
+
+            if (connection != null) {
+                try {
+                    connection.close();
+                } catch (Exception ex) {
+
+                }
+            }
+        }
+        return numberConsumedMessage;
+    }
+
+}
+