resolve https://issues.apache.org/activemq/browse/AMQ-1112 - fix issues with inflight count when messages expire on the consumer/client and on consumer close

git-svn-id: https://svn.apache.org/repos/asf/activemq/trunk@791881 13f79535-47bb-0310-9956-ffa450edef68
This commit is contained in:
Gary Tully 2009-07-07 15:54:32 +00:00
parent 8c70e4a780
commit 7a59d284dc
9 changed files with 219 additions and 77 deletions

View File

@ -644,6 +644,9 @@ public class ActiveMQMessageConsumer implements MessageAvailableConsumer, StatsC
if (ack != null) { if (ack != null) {
deliveredMessages.clear(); deliveredMessages.clear();
ackCounter = 0; ackCounter = 0;
} else {
ack = pendingAck;
pendingAck = null;
} }
} }
} else if (pendingAck != null && pendingAck.isStandardAck()) { } else if (pendingAck != null && pendingAck.isStandardAck()) {

View File

@ -69,7 +69,7 @@ public class IndirectMessageReference implements QueueMessageReference {
} }
public String toString() { public String toString() {
return "Message " + message.getMessageId() + " dropped=" + dropped + " locked=" + (lockOwner != null); return "Message " + message.getMessageId() + " dropped=" + dropped + " acked=" + acked + " locked=" + (lockOwner != null);
} }
public void incrementRedeliveryCounter() { public void incrementRedeliveryCounter() {

View File

@ -303,9 +303,16 @@ public abstract class PrefetchSubscription extends AbstractSubscription {
int index = 0; int index = 0;
for (Iterator<MessageReference> iter = dispatched.iterator(); iter.hasNext(); index++) { for (Iterator<MessageReference> iter = dispatched.iterator(); iter.hasNext(); index++) {
final MessageReference node = iter.next(); final MessageReference node = iter.next();
if( node.isExpired() ) { if (hasNotAlreadyExpired(node)) {
node.getRegionDestination().messageExpired(context, this, node); if (node.isExpired()) {
node.getRegionDestination().messageExpired(context, this, node);
dispatched.remove(node);
node.getRegionDestination().getDestinationStatistics().getInflight().decrement();
}
} else {
// already expired
dispatched.remove(node); dispatched.remove(node);
node.getRegionDestination().getDestinationStatistics().getInflight().decrement();
} }
if (ack.getLastMessageId().equals(node.getMessageId())) { if (ack.getLastMessageId().equals(node.getMessageId())) {
prefetchExtension = Math.max(prefetchExtension, index + 1); prefetchExtension = Math.max(prefetchExtension, index + 1);
@ -411,6 +418,16 @@ public abstract class PrefetchSubscription extends AbstractSubscription {
} }
} }
private boolean hasNotAlreadyExpired(MessageReference node) {
boolean hasNotExpired = true;
try {
hasNotExpired = node.getMessage().getProperty(RegionBroker.ORIGINAL_EXPIRATION) == null;
} catch (IOException e) {
LOG.warn("failed to determine value message property " + RegionBroker.ORIGINAL_EXPIRATION + " for " + node, e);
}
return hasNotExpired;
}
/** /**
* Checks an ack versus the contents of the dispatched list. * Checks an ack versus the contents of the dispatched list.
* *
@ -545,6 +562,11 @@ public abstract class PrefetchSubscription extends AbstractSubscription {
List<MessageReference> rc = new ArrayList<MessageReference>(); List<MessageReference> rc = new ArrayList<MessageReference>();
synchronized(pendingLock) { synchronized(pendingLock) {
super.remove(context, destination); super.remove(context, destination);
// Here is a potential problem concerning Inflight stat:
// Messages not already committed or rolled back may not be removed from dispatched list at the moment
// Except if each commit or rollback callback action comes before remove of subscriber.
rc.addAll(pending.remove(context, destination));
// Synchronized to DispatchLock // Synchronized to DispatchLock
synchronized(dispatchLock) { synchronized(dispatchLock) {
for (MessageReference r : dispatched) { for (MessageReference r : dispatched) {
@ -552,12 +574,10 @@ public abstract class PrefetchSubscription extends AbstractSubscription {
rc.add((QueueMessageReference)r); rc.add((QueueMessageReference)r);
} }
} }
} destination.getDestinationStatistics().getDispatched().subtract(dispatched.size());
// TODO Dispatched messages should be decremented from Inflight stat destination.getDestinationStatistics().getInflight().subtract(dispatched.size());
// Here is a potential problem concerning Inflight stat: dispatched.clear();
// Messages not already committed or rolled back may not be removed from dispatched list at the moment }
// Except if each commit or rollback callback action comes before remove of subscriber.
rc.addAll(pending.remove(context, destination));
} }
return rc; return rc;
} }
@ -661,12 +681,15 @@ public abstract class PrefetchSubscription extends AbstractSubscription {
if (node.getRegionDestination() != null) { if (node.getRegionDestination() != null) {
if (node != QueueMessageReference.NULL_MESSAGE) { if (node != QueueMessageReference.NULL_MESSAGE) {
node.getRegionDestination().getDestinationStatistics().getDispatched().increment(); node.getRegionDestination().getDestinationStatistics().getDispatched().increment();
node.getRegionDestination().getDestinationStatistics().getInflight().increment(); node.getRegionDestination().getDestinationStatistics().getInflight().increment();
if (LOG.isTraceEnabled()) {
LOG.trace(info.getDestination().getPhysicalName() + " dispatched: " + message.getMessageId()
+ ", dispatched: " + node.getRegionDestination().getDestinationStatistics().getDispatched().getCount()
+ ", inflight: " + node.getRegionDestination().getDestinationStatistics().getInflight().getCount());
}
} }
} }
if (LOG.isTraceEnabled()) {
LOG.trace(info.getDestination().getPhysicalName() + " dispatched: " + message.getMessageId());
}
if (info.isDispatchAsync()) { if (info.isDispatchAsync()) {
try { try {
dispatchPending(); dispatchPending();

View File

@ -205,7 +205,7 @@ public class Queue extends BaseDestination implements Task, UsageListener {
// Message could have expired while it was being // Message could have expired while it was being
// loaded.. // loaded..
if (broker.isExpired(message)) { if (broker.isExpired(message)) {
messageExpired(createConnectionContext(), null, message, false); messageExpired(createConnectionContext(), message);
return true; return true;
} }
if (hasSpace()) { if (hasSpace()) {
@ -343,6 +343,12 @@ public class Queue extends BaseDestination implements Task, UsageListener {
// while removing up a subscription. // while removing up a subscription.
dispatchLock.lock(); dispatchLock.lock();
try { try {
if (LOG.isDebugEnabled()) {
LOG.debug("remove sub: " + sub + ", lastDeliveredSeqId: " + lastDeiveredSequenceId
+ ", dequeues: " + getDestinationStatistics().getDequeues().getCount()
+ ", dispatched: " + getDestinationStatistics().getDispatched().getCount()
+ ", inflight: " + getDestinationStatistics().getInflight().getCount());
}
synchronized (consumers) { synchronized (consumers) {
removeFromConsumerList(sub); removeFromConsumerList(sub);
if (sub.getConsumerInfo().isExclusive()) { if (sub.getConsumerInfo().isExclusive()) {
@ -552,10 +558,12 @@ public class Queue extends BaseDestination implements Task, UsageListener {
} }
private void expireMessages() { private void expireMessages() {
LOG.info("expiring messages..."); if (LOG.isDebugEnabled()) {
LOG.debug("Expiring messages ..");
}
// just track the insertion count // just track the insertion count
List<Message> l = new AbstractList<Message>() { List<Message> browsedMessages = new AbstractList<Message>() {
int size = 0; int size = 0;
@Override @Override
@ -573,7 +581,7 @@ public class Queue extends BaseDestination implements Task, UsageListener {
return null; return null;
} }
}; };
doBrowse(true, l, getMaxBrowsePageSize()); doBrowse(true, browsedMessages, this.getMaxExpirePageSize());
} }
public void gc(){ public void gc(){
@ -750,7 +758,7 @@ public class Queue extends BaseDestination implements Task, UsageListener {
addAll(pagedInPendingDispatch, l, max, toExpire); addAll(pagedInPendingDispatch, l, max, toExpire);
for (MessageReference ref : toExpire) { for (MessageReference ref : toExpire) {
pagedInPendingDispatch.remove(ref); pagedInPendingDispatch.remove(ref);
messageExpired(connectionContext, ref, false); messageExpired(connectionContext, ref);
} }
} }
toExpire.clear(); toExpire.clear();
@ -758,7 +766,7 @@ public class Queue extends BaseDestination implements Task, UsageListener {
addAll(pagedInMessages.values(), l, max, toExpire); addAll(pagedInMessages.values(), l, max, toExpire);
} }
for (MessageReference ref : toExpire) { for (MessageReference ref : toExpire) {
messageExpired(connectionContext, ref, false); messageExpired(connectionContext, ref);
} }
if (l.size() < getMaxBrowsePageSize()) { if (l.size() < getMaxBrowsePageSize()) {
@ -771,7 +779,7 @@ public class Queue extends BaseDestination implements Task, UsageListener {
if (node != null) { if (node != null) {
if (broker.isExpired(node)) { if (broker.isExpired(node)) {
messageExpired(connectionContext, messageExpired(connectionContext,
createMessageReference(node.getMessage()), false); createMessageReference(node.getMessage()));
} else if (l.contains(node.getMessage()) == false) { } else if (l.contains(node.getMessage()) == false) {
l.add(node.getMessage()); l.add(node.getMessage());
} }
@ -1249,21 +1257,17 @@ public class Queue extends BaseDestination implements Task, UsageListener {
} }
} }
public void messageExpired(ConnectionContext context,MessageReference reference, boolean dispatched) { public void messageExpired(ConnectionContext context,MessageReference reference) {
messageExpired(context,null,reference, dispatched); messageExpired(context,null,reference);
} }
public void messageExpired(ConnectionContext context,Subscription subs, MessageReference reference) { public void messageExpired(ConnectionContext context,Subscription subs, MessageReference reference) {
messageExpired(context, subs, reference, true); if (LOG.isDebugEnabled()) {
} LOG.debug("message expired: " + reference);
}
public void messageExpired(ConnectionContext context,Subscription subs, MessageReference reference, boolean dispatched) {
broker.messageExpired(context, reference); broker.messageExpired(context, reference);
destinationStatistics.getDequeues().increment(); destinationStatistics.getDequeues().increment();
destinationStatistics.getExpired().increment(); destinationStatistics.getExpired().increment();
if (dispatched) {
destinationStatistics.getInflight().decrement();
}
try { try {
removeMessage(context,subs,(QueueMessageReference)reference); removeMessage(context,subs,(QueueMessageReference)reference);
} catch (IOException e) { } catch (IOException e) {
@ -1349,7 +1353,7 @@ public class Queue extends BaseDestination implements Task, UsageListener {
result.add(ref); result.add(ref);
count++; count++;
} else { } else {
messageExpired(createConnectionContext(), ref, false); messageExpired(createConnectionContext(), ref);
} }
} }
} finally { } finally {

View File

@ -69,6 +69,7 @@ import org.apache.commons.logging.LogFactory;
* @version $Revision$ * @version $Revision$
*/ */
public class RegionBroker extends EmptyBroker { public class RegionBroker extends EmptyBroker {
public static final String ORIGINAL_EXPIRATION = "originalExpiration";
private static final Log LOG = LogFactory.getLog(RegionBroker.class); private static final Log LOG = LogFactory.getLog(RegionBroker.class);
private static final IdGenerator BROKER_ID_GENERATOR = new IdGenerator(); private static final IdGenerator BROKER_ID_GENERATOR = new IdGenerator();
@ -689,7 +690,7 @@ public class RegionBroker extends EmptyBroker {
} }
long expiration=message.getExpiration(); long expiration=message.getExpiration();
message.setExpiration(0); message.setExpiration(0);
message.setProperty("originalExpiration",new Long( message.setProperty(ORIGINAL_EXPIRATION,new Long(
expiration)); expiration));
if(!message.isPersistent()){ if(!message.isPersistent()){
message.setPersistent(true); message.setPersistent(true);

View File

@ -277,7 +277,7 @@ public class DurableConsumerTest extends TestCase {
executor.shutdown(); executor.shutdown();
executor.awaitTermination(30, TimeUnit.SECONDS); executor.awaitTermination(30, TimeUnit.SECONDS);
assertTrue("got some messages: " + receivedCount.get(), receivedCount.get() > numMessages); assertTrue("got some messages: " + receivedCount.get(), receivedCount.get() > numMessages);
assertTrue(exceptions.isEmpty()); assertTrue("no exceptions, but: " + exceptions, exceptions.isEmpty());
} }
public void testConsumerRecover() throws Exception { public void testConsumerRecover() throws Exception {

View File

@ -354,12 +354,7 @@ public class ActiveMQMapMessageTest extends TestCase {
mapMessage.onSend(); mapMessage.onSend();
mapMessage.setContent(mapMessage.getContent()); mapMessage.setContent(mapMessage.getContent());
try { assertNull(mapMessage.getString("String"));
mapMessage.getString("String");
fail("Should throw a Null pointer");
}catch(NullPointerException e){
}
mapMessage.clearBody(); mapMessage.clearBody();
mapMessage.setString("String", "String"); mapMessage.setString("String", "String");

View File

@ -32,14 +32,13 @@ import org.apache.activemq.ActiveMQConnectionFactory;
import org.apache.activemq.CombinationTestSupport; import org.apache.activemq.CombinationTestSupport;
import org.apache.activemq.broker.BrokerService; import org.apache.activemq.broker.BrokerService;
import org.apache.activemq.broker.jmx.DestinationViewMBean; import org.apache.activemq.broker.jmx.DestinationViewMBean;
import org.apache.activemq.broker.region.policy.PolicyEntry;
import org.apache.activemq.broker.region.policy.PolicyMap;
import org.apache.activemq.command.ActiveMQDestination; import org.apache.activemq.command.ActiveMQDestination;
import org.apache.activemq.command.ActiveMQQueue; import org.apache.activemq.command.ActiveMQQueue;
import org.apache.activemq.command.ActiveMQTopic;
import org.apache.commons.logging.Log; import org.apache.commons.logging.Log;
import org.apache.commons.logging.LogFactory; import org.apache.commons.logging.LogFactory;
public class ExpiredMessagesTest extends CombinationTestSupport { public class ExpiredMessagesTest extends CombinationTestSupport {
private static final Log LOG = LogFactory.getLog(ExpiredMessagesTest.class); private static final Log LOG = LogFactory.getLog(ExpiredMessagesTest.class);
@ -60,15 +59,22 @@ public class ExpiredMessagesTest extends CombinationTestSupport {
} }
protected void setUp() throws Exception { protected void setUp() throws Exception {
broker = new BrokerService(); broker = new BrokerService();
broker.setBrokerName("localhost"); broker.setBrokerName("localhost");
broker.setDataDirectory("data/"); broker.setDataDirectory("data/");
broker.setUseJmx(true); broker.setUseJmx(true);
broker.deleteAllMessages(); broker.deleteAllMessages();
broker.addConnector("tcp://localhost:61616");
broker.start(); PolicyEntry defaultPolicy = new PolicyEntry();
broker.waitUntilStarted(); defaultPolicy.setExpireMessagesPeriod(100);
} PolicyMap policyMap = new PolicyMap();
policyMap.setDefaultEntry(defaultPolicy);
broker.setDestinationPolicy(policyMap);
broker.addConnector("tcp://localhost:61616");
broker.start();
broker.waitUntilStarted();
}
public void testExpiredMessages() throws Exception { public void testExpiredMessages() throws Exception {
@ -93,6 +99,7 @@ public class ExpiredMessagesTest extends CombinationTestSupport {
Thread.sleep(100); Thread.sleep(100);
end = System.currentTimeMillis(); end = System.currentTimeMillis();
} }
consumer.close();
} catch (Throwable ex) { } catch (Throwable ex) {
ex.printStackTrace(); ex.printStackTrace();
} }
@ -109,6 +116,7 @@ public class ExpiredMessagesTest extends CombinationTestSupport {
while (i++ < 30000) { while (i++ < 30000) {
producer.send(session.createTextMessage("test")); producer.send(session.createTextMessage("test"));
} }
producer.close();
} catch (Throwable ex) { } catch (Throwable ex) {
ex.printStackTrace(); ex.printStackTrace();
} }
@ -119,14 +127,23 @@ public class ExpiredMessagesTest extends CombinationTestSupport {
consumerThread.join(); consumerThread.join();
producingThread.join(); producingThread.join();
session.close();
Thread.sleep(5000);
DestinationViewMBean view = createView(destination); DestinationViewMBean view = createView(destination);
LOG.info("Stats: received: " + received.get() + ", enqueues: " + view.getDequeueCount() + ", dequeues: " + view.getDequeueCount() LOG.info("Stats: received: " + received.get() + ", enqueues: " + view.getDequeueCount() + ", dequeues: " + view.getDequeueCount()
+ ", dispatched: " + view.getDispatchCount() + ", inflight: " + view.getInFlightCount() + ", expiries: " + view.getExpiredCount()); + ", dispatched: " + view.getDispatchCount() + ", inflight: " + view.getInFlightCount() + ", expiries: " + view.getExpiredCount());
assertEquals("got what did not expire", received.get(), view.getDequeueCount() - view.getExpiredCount()); assertEquals("got what did not expire", received.get(), view.getDequeueCount() - view.getExpiredCount());
//assertEquals("Wrong inFlightCount: " + view.getInFlightCount(), view.getDispatchCount() - view.getDequeueCount(), view.getInFlightCount());
long expiry = System.currentTimeMillis() + 30000;
while (view.getInFlightCount() > 0 && System.currentTimeMillis() < expiry) {
Thread.sleep(500);
}
LOG.info("Stats: received: " + received.get() + ", enqueues: " + view.getDequeueCount() + ", dequeues: " + view.getDequeueCount()
+ ", dispatched: " + view.getDispatchCount() + ", inflight: " + view.getInFlightCount() + ", expiries: " + view.getExpiredCount());
assertEquals("Wrong inFlightCount: ", 0, view.getInFlightCount());
} }
protected DestinationViewMBean createView(ActiveMQDestination destination) throws Exception { protected DestinationViewMBean createView(ActiveMQDestination destination) throws Exception {
@ -146,7 +163,4 @@ public class ExpiredMessagesTest extends CombinationTestSupport {
broker.stop(); broker.stop();
broker.waitUntilStopped(); broker.waitUntilStopped();
} }
} }

View File

@ -16,7 +16,13 @@
*/ */
package org.apache.activemq.usecases; package org.apache.activemq.usecases;
import java.util.concurrent.CountDownLatch;
import java.util.concurrent.TimeUnit;
import javax.jms.Connection; import javax.jms.Connection;
import javax.jms.Message;
import javax.jms.MessageConsumer;
import javax.jms.MessageListener;
import javax.jms.MessageProducer; import javax.jms.MessageProducer;
import javax.jms.Session; import javax.jms.Session;
import javax.management.MBeanServer; import javax.management.MBeanServer;
@ -57,31 +63,45 @@ public class ExpiredMessagesWithNoConsumerTest extends CombinationTestSupport {
junit.textui.TestRunner.run(suite()); junit.textui.TestRunner.run(suite());
} }
protected void setUp() throws Exception { protected void createBrokerWithMemoryLimit() throws Exception {
broker = new BrokerService(); doCreateBroker(true);
broker.setBrokerName("localhost"); }
broker.setDataDirectory("data/");
broker.setUseJmx(true); protected void createBroker() throws Exception {
broker.setDeleteAllMessagesOnStartup(true); doCreateBroker(false);
broker.addConnector("tcp://localhost:61616"); }
PolicyMap policyMap = new PolicyMap(); private void doCreateBroker(boolean memoryLimit) throws Exception {
PolicyEntry defaultEntry = new PolicyEntry(); broker = new BrokerService();
defaultEntry.setExpireMessagesPeriod(expiryPeriod); broker.setBrokerName("localhost");
defaultEntry.setMaxExpirePageSize(200); broker.setDataDirectory("data/");
// so memory is not consumed by DLQ turn if off broker.setUseJmx(true);
defaultEntry.setDeadLetterStrategy(null); broker.setDeleteAllMessagesOnStartup(true);
defaultEntry.setMemoryLimit(200*1000); broker.addConnector("tcp://localhost:61616");
policyMap.setDefaultEntry(defaultEntry);
PolicyMap policyMap = new PolicyMap();
PolicyEntry defaultEntry = new PolicyEntry();
defaultEntry.setExpireMessagesPeriod(expiryPeriod);
defaultEntry.setMaxExpirePageSize(200);
if (memoryLimit) {
// so memory is not consumed by DLQ turn if off
defaultEntry.setDeadLetterStrategy(null);
defaultEntry.setMemoryLimit(200 * 1000);
}
policyMap.setDefaultEntry(defaultEntry);
broker.setDestinationPolicy(policyMap); broker.setDestinationPolicy(policyMap);
broker.start();
broker.waitUntilStarted();
}
broker.start(); public void testExpiredMessagesWithNoConsumer() throws Exception {
broker.waitUntilStarted();
}
public void testExpiredMessages() throws Exception {
createBrokerWithMemoryLimit();
ActiveMQConnectionFactory factory = new ActiveMQConnectionFactory("tcp://localhost:61616"); ActiveMQConnectionFactory factory = new ActiveMQConnectionFactory("tcp://localhost:61616");
connection = factory.createConnection(); connection = factory.createConnection();
session = connection.createSession(false, Session.AUTO_ACKNOWLEDGE); session = connection.createSession(false, Session.AUTO_ACKNOWLEDGE);
@ -121,7 +141,89 @@ public class ExpiredMessagesWithNoConsumerTest extends CombinationTestSupport {
DestinationViewMBean view = createView(destination); DestinationViewMBean view = createView(destination);
assertEquals("All sent have expired ", sendCount, view.getExpiredCount()); assertEquals("All sent have expired ", sendCount, view.getExpiredCount());
} }
public void testExpiredMessagesWitVerySlowConsumer() throws Exception {
createBroker();
ActiveMQConnectionFactory factory = new ActiveMQConnectionFactory("tcp://localhost:61616");
connection = factory.createConnection();
session = connection.createSession(false, Session.CLIENT_ACKNOWLEDGE);
producer = session.createProducer(destination);
final int ttl = 4000;
producer.setTimeToLive(ttl);
final long sendCount = 1001;
final CountDownLatch receivedOneCondition = new CountDownLatch(1);
final CountDownLatch waitCondition = new CountDownLatch(1);
MessageConsumer consumer = session.createConsumer(destination);
consumer.setMessageListener(new MessageListener() {
public void onMessage(Message message) {
try {
LOG.info("Got my message: " + message);
receivedOneCondition.countDown();
waitCondition.await(60, TimeUnit.SECONDS);
message.acknowledge();
} catch (Exception e) {
e.printStackTrace();
fail(e.toString());
}
}
});
connection.start();
Thread producingThread = new Thread("Producing Thread") {
public void run() {
try {
int i = 0;
long tStamp = System.currentTimeMillis();
while (i++ < sendCount) {
producer.send(session.createTextMessage("test"));
if (i%100 == 0) {
LOG.info("sent: " + i + " @ " + ((System.currentTimeMillis() - tStamp) / 100) + "m/ms");
tStamp = System.currentTimeMillis() ;
}
}
} catch (Throwable ex) {
ex.printStackTrace();
}
}
};
producingThread.start();
final long expiry = System.currentTimeMillis() + 20*1000;
while (producingThread.isAlive() && expiry > System.currentTimeMillis()) {
producingThread.join(1000);
}
assertTrue("got one message", receivedOneCondition.await(10, TimeUnit.SECONDS));
assertTrue("producer completed within time ", !producingThread.isAlive());
Thread.sleep(2 * Math.max(ttl, expiryPeriod));
DestinationViewMBean view = createView(destination);
assertEquals("all dispatched up to default prefetch ", 1000, view.getDispatchCount());
assertEquals("All sent save one have expired ", sendCount, view.getExpiredCount());
// let the ack happen
waitCondition.countDown();
Thread.sleep(Math.max(ttl, expiryPeriod));
assertEquals("all sent save one have expired ", sendCount, view.getExpiredCount());
assertEquals("prefetch gets back to 0 ", 0, view.getInFlightCount());
consumer.close();
LOG.info("done: " + getName());
}
protected DestinationViewMBean createView(ActiveMQDestination destination) throws Exception { protected DestinationViewMBean createView(ActiveMQDestination destination) throws Exception {
MBeanServer mbeanServer = broker.getManagementContext().getMBeanServer(); MBeanServer mbeanServer = broker.getManagementContext().getMBeanServer();
String domain = "org.apache.activemq"; String domain = "org.apache.activemq";