ARTEMIS-3848 High CPU usage on ReadWriteLocks

This is caused by too many entries on the HashMap for ThreadLocals.
Also: I'm reviewing some readlock usage on the StorageManager to simplify things a little bit.
This commit is contained in:
Clebert Suconic 2022-05-25 09:21:51 -04:00 committed by clebertsuconic
parent f14ba5f742
commit a2dd805a2f
13 changed files with 122 additions and 170 deletions

View File

@ -25,27 +25,23 @@ package org.apache.activemq.artemis.utils.actors;
*/ */
public abstract class HandlerBase { public abstract class HandlerBase {
//marker instance used to recognize if a thread is performing a packet handling private static class Counter {
private static final Object DUMMY = Boolean.TRUE; int count = 0;
// this cannot be static as the Actor will be used within another executor. For that reason
// each instance will have its own ThreadLocal.
// ... a thread that has its thread-local map populated with DUMMY while performing a handler
private final ThreadLocal<Object> inHandler = new ThreadLocal<>();
protected void enter() {
assert inHandler.get() == null : "should be null";
inHandler.set(DUMMY);
} }
public boolean inHandler() { /** an actor could be used within an OrderedExecutor. So we need this counter to decide if there's a Handler anywhere in the stack trace */
final Object dummy = inHandler.get(); private static final ThreadLocal<Counter> inHandler = ThreadLocal.withInitial(() -> new Counter());
return dummy != null;
protected static void enter() {
inHandler.get().count++;
} }
protected void leave() { public static boolean inHandler() {
assert inHandler.get() != null : "marker not set"; return inHandler.get().count > 0;
inHandler.set(null); }
protected static void leave() {
inHandler.get().count--;
} }
} }

View File

@ -62,7 +62,7 @@ public class NettyConnection implements Connection {
* here for when the connection (or Netty Channel) becomes available again. * here for when the connection (or Netty Channel) becomes available again.
*/ */
private final List<ReadyListener> readyListeners = new ArrayList<>(); private final List<ReadyListener> readyListeners = new ArrayList<>();
private final FastThreadLocal<ArrayList<ReadyListener>> localListenersPool = new FastThreadLocal<>(); private static final FastThreadLocal<ArrayList<ReadyListener>> readyListenersPool = new FastThreadLocal<>();
private final boolean batchingEnabled; private final boolean batchingEnabled;
@ -138,37 +138,37 @@ public class NettyConnection implements Connection {
@Override @Override
public final void fireReady(final boolean ready) { public final void fireReady(final boolean ready) {
ArrayList<ReadyListener> readyToCall = localListenersPool.get();
if (readyToCall != null) { // We are reusing a previously created ArrayList for this localArray
localListenersPool.set(null); ArrayList<ReadyListener> localArrayList = readyListenersPool.get();
if (localArrayList == null) {
localArrayList = new ArrayList<>();
readyListenersPool.set(localArrayList);
} else {
localArrayList.clear();
} }
synchronized (readyListeners) { synchronized (readyListeners) {
this.ready = ready; this.ready = ready;
if (ready) { if (ready) {
final int size = this.readyListeners.size(); final int size = this.readyListeners.size();
if (readyToCall != null) { localArrayList.ensureCapacity(size);
readyToCall.ensureCapacity(size);
}
try { try {
for (int i = 0; i < size; i++) { for (int i = 0; i < size; i++) {
final ReadyListener readyListener = readyListeners.get(i); final ReadyListener readyListener = readyListeners.get(i);
if (readyListener == null) { if (readyListener == null) {
break; break;
} }
if (readyToCall == null) { localArrayList.add(readyListener);
readyToCall = new ArrayList<>(size);
}
readyToCall.add(readyListener);
} }
} finally { } finally {
readyListeners.clear(); readyListeners.clear();
} }
} }
} }
if (readyToCall != null) {
try { try {
readyToCall.forEach(readyListener -> { localArrayList.forEach(readyListener -> {
try { try {
readyListener.readyForWriting(); readyListener.readyForWriting();
} catch (Throwable logOnly) { } catch (Throwable logOnly) {
@ -178,11 +178,7 @@ public class NettyConnection implements Connection {
} catch (Throwable t) { } catch (Throwable t) {
ActiveMQClientLogger.LOGGER.failedToSetChannelReadyForWriting(t); ActiveMQClientLogger.LOGGER.failedToSetChannelReadyForWriting(t);
} finally { } finally {
readyToCall.clear(); localArrayList.clear();
if (localListenersPool.get() != null) {
localListenersPool.set(readyToCall);
}
}
} }
} }

View File

@ -18,7 +18,6 @@ package org.apache.activemq.artemis.core.paging;
import java.io.File; import java.io.File;
import java.util.Collection; import java.util.Collection;
import java.util.concurrent.locks.ReentrantReadWriteLock.ReadLock;
import org.apache.activemq.artemis.api.core.Message; import org.apache.activemq.artemis.api.core.Message;
import org.apache.activemq.artemis.api.core.RefCountMessageListener; import org.apache.activemq.artemis.api.core.RefCountMessageListener;
@ -89,14 +88,11 @@ public interface PagingStore extends ActiveMQComponent, RefCountMessageListener
/** /**
* Write message to page if we are paging. * Write message to page if we are paging.
* *
* @param readLock a read lock from the storage manager. This is an encapsulation violation made
* to keep the code less complex. If give {@code null} the method will throw a
* {@link NullPointerException}
* @return {@code true} if we are paging and have handled the data, {@code false} if the data * @return {@code true} if we are paging and have handled the data, {@code false} if the data
* needs to be sent to the journal * needs to be sent to the journal
* @throws NullPointerException if {@code readLock} is null * @throws NullPointerException if {@code readLock} is null
*/ */
boolean page(Message message, Transaction tx, RouteContextList listCtx, ReadLock readLock) throws Exception; boolean page(Message message, Transaction tx, RouteContextList listCtx) throws Exception;
Page createPage(int page) throws Exception; Page createPage(int page) throws Exception;

View File

@ -30,7 +30,6 @@ import java.util.concurrent.ScheduledExecutorService;
import java.util.concurrent.TimeUnit; import java.util.concurrent.TimeUnit;
import java.util.concurrent.locks.ReadWriteLock; import java.util.concurrent.locks.ReadWriteLock;
import java.util.concurrent.locks.ReentrantReadWriteLock; import java.util.concurrent.locks.ReentrantReadWriteLock;
import java.util.concurrent.locks.ReentrantReadWriteLock.ReadLock;
import org.apache.activemq.artemis.api.core.Message; import org.apache.activemq.artemis.api.core.Message;
import org.apache.activemq.artemis.api.core.SimpleString; import org.apache.activemq.artemis.api.core.SimpleString;
@ -897,8 +896,7 @@ public class PagingStoreImpl implements PagingStore {
@Override @Override
public boolean page(Message message, public boolean page(Message message,
final Transaction tx, final Transaction tx,
RouteContextList listCtx, RouteContextList listCtx) throws Exception {
final ReadLock managerLock) throws Exception {
if (!running) { if (!running) {
return false; return false;
@ -941,10 +939,6 @@ public class PagingStoreImpl implements PagingStore {
lock.readLock().unlock(); lock.readLock().unlock();
} }
if (managerLock != null) {
managerLock.lock();
}
try {
lock.writeLock().lock(); lock.writeLock().lock();
try { try {
@ -993,11 +987,6 @@ public class PagingStoreImpl implements PagingStore {
} finally { } finally {
lock.writeLock().unlock(); lock.writeLock().unlock();
} }
} finally {
if (managerLock != null) {
managerLock.unlock();
}
}
} }
/** /**

View File

@ -436,11 +436,6 @@ public interface StorageManager extends IDGenerator, ActiveMQComponent {
/** /**
* Write message to page if we are paging. * Write message to page if we are paging.
* <p>
* This is primarily a {@link PagingStore} call, but as with any other call writing persistent
* data, it must go through here. Both for the sake of replication, and also to ensure that it
* takes the locks (storage manager and pagingStore) in the right order. Avoiding thus the
* creation of dead-locks.
* *
* @return {@code true} if we are paging and have handled the data, {@code false} if the data * @return {@code true} if we are paging and have handled the data, {@code false} if the data
* needs to be sent to the journal * needs to be sent to the journal

View File

@ -174,12 +174,20 @@ public abstract class AbstractJournalStorageManager extends CriticalComponentImp
protected final ScheduledExecutorService scheduledExecutorService; protected final ScheduledExecutorService scheduledExecutorService;
protected final ReentrantReadWriteLock storageManagerLock = new ReentrantReadWriteLock(true); protected final ReentrantReadWriteLock storageManagerLock = new ReentrantReadWriteLock(false);
// I would rather cache the Closeable instance here.. // I would rather cache the Closeable instance here..
// I never know when the JRE decides to create a new instance on every call. // I never know when the JRE decides to create a new instance on every call.
// So I'm playing safe here. That's all // So I'm playing safe here. That's all
protected final ArtemisCloseable unlockCloseable = storageManagerLock.readLock()::unlock; protected final ArtemisCloseable unlockCloseable = this::unlockCloseable;
protected static final ArtemisCloseable dummyCloseable = () -> { };
private static final ThreadLocal<Boolean> reentrant = ThreadLocal.withInitial(() -> false);
private void unlockCloseable() {
storageManagerLock.readLock().unlock();
reentrant.set(false);
}
protected Journal messageJournal; protected Journal messageJournal;
@ -395,6 +403,12 @@ public abstract class AbstractJournalStorageManager extends CriticalComponentImp
@Override @Override
public ArtemisCloseable closeableReadLock() { public ArtemisCloseable closeableReadLock() {
if (reentrant.get()) {
return dummyCloseable;
}
reentrant.set(true);
CriticalCloseable measure = measureCritical(CRITICAL_STORE); CriticalCloseable measure = measureCritical(CRITICAL_STORE);
storageManagerLock.readLock().lock(); storageManagerLock.readLock().lock();
@ -415,20 +429,6 @@ public abstract class AbstractJournalStorageManager extends CriticalComponentImp
} }
} }
/**
* for internal use and testsuite, don't use it outside of tests
*/
public void writeLock() {
storageManagerLock.writeLock().lock();
}
/**
* for internal use and testsuite, don't use it outside of tests
*/
public void writeUnlock() {
storageManagerLock.writeLock().unlock();
}
@Override @Override
public void storeAcknowledge(final long queueID, final long messageID) throws Exception { public void storeAcknowledge(final long queueID, final long messageID) throws Exception {
try (ArtemisCloseable lock = closeableReadLock()) { try (ArtemisCloseable lock = closeableReadLock()) {
@ -2150,17 +2150,9 @@ public abstract class AbstractJournalStorageManager extends CriticalComponentImp
@Override @Override
public boolean addToPage(PagingStore store, Message msg, Transaction tx, RouteContextList listCtx) throws Exception { public boolean addToPage(PagingStore store, Message msg, Transaction tx, RouteContextList listCtx) throws Exception {
/** try (ArtemisCloseable closeable = closeableReadLock()) {
* Exposing the read-lock here is an encapsulation violation done in order to keep the code return store.page(msg, tx, listCtx);
* simpler. The alternative would be to add a second method, say 'verifyPaging', to }
* PagingStore.
* <p>
* Adding this second method would also be more surprise prone as it would require a certain
* calling order.
* <p>
* The reasoning is that exposing the lock is more explicit and therefore `less bad`.
*/
return store.page(msg, tx, listCtx, storageManagerLock.readLock());
} }
private void installLargeMessageConfirmationOnTX(Transaction tx, long recordID) { private void installLargeMessageConfirmationOnTX(Transaction tx, long recordID) {

View File

@ -642,18 +642,8 @@ public class NullStorageManager implements StorageManager {
Message msg, Message msg,
Transaction tx, Transaction tx,
RouteContextList listCtx) throws Exception { RouteContextList listCtx) throws Exception {
/**
* Exposing the read-lock here is an encapsulation violation done in order to keep the code
* simpler. The alternative would be to add a second method, say 'verifyPaging', to
* PagingStore.
* <p>
* Adding this second method would also be more surprise prone as it would require a certain
* calling order.
* <p>
* The reasoning is that exposing the lock is more explicit and therefore `less bad`.
*/
if (store != null) { if (store != null) {
return store.page(msg, tx, listCtx, null); return store.page(msg, tx, listCtx);
} else { } else {
return false; return false;
} }

View File

@ -1131,7 +1131,7 @@
<arg>--add-exports=jdk.unsupported/sun.misc=ALL-UNNAMED</arg> <arg>--add-exports=jdk.unsupported/sun.misc=ALL-UNNAMED</arg>
<arg>--add-exports=java.base/jdk.internal.misc=ALL-UNNAMED</arg> <arg>--add-exports=java.base/jdk.internal.misc=ALL-UNNAMED</arg>
<arg>-XDcompilePolicy=simple</arg> <arg>-XDcompilePolicy=simple</arg>
<arg>-Xplugin:ErrorProne -Xep:MissingOverride:ERROR -Xep:NonAtomicVolatileUpdate:ERROR -Xep:SynchronizeOnNonFinalField:ERROR -Xep:StaticQualifiedUsingExpression:ERROR -Xep:WaitNotInLoop:ERROR -XepExcludedPaths:.*/generated-sources/.*</arg> <arg>-Xplugin:ErrorProne -Xep:ThreadLocalUsage:ERROR -Xep:MissingOverride:ERROR -Xep:NonAtomicVolatileUpdate:ERROR -Xep:SynchronizeOnNonFinalField:ERROR -Xep:StaticQualifiedUsingExpression:ERROR -Xep:WaitNotInLoop:ERROR -XepExcludedPaths:.*/generated-sources/.*</arg>
</compilerArgs> </compilerArgs>
</configuration> </configuration>
</plugin> </plugin>
@ -1165,7 +1165,7 @@
<arg>--add-exports=jdk.unsupported/sun.misc=ALL-UNNAMED</arg> <arg>--add-exports=jdk.unsupported/sun.misc=ALL-UNNAMED</arg>
<arg>--add-exports=java.base/jdk.internal.misc=ALL-UNNAMED</arg> <arg>--add-exports=java.base/jdk.internal.misc=ALL-UNNAMED</arg>
<arg>-XDcompilePolicy=simple</arg> <arg>-XDcompilePolicy=simple</arg>
<arg>-Xplugin:ErrorProne -Xep:MissingOverride:WARN -Xep:NonAtomicVolatileUpdate:ERROR -Xep:SynchronizeOnNonFinalField:ERROR -Xep:StaticQualifiedUsingExpression:ERROR -Xep:WaitNotInLoop:ERROR -XepExcludedPaths:.*/generated-sources/.*</arg> <arg>-Xplugin:ErrorProne -Xep:ThreadLocalUsage:ERROR -Xep:MissingOverride:WARN -Xep:NonAtomicVolatileUpdate:ERROR -Xep:SynchronizeOnNonFinalField:ERROR -Xep:StaticQualifiedUsingExpression:ERROR -Xep:WaitNotInLoop:ERROR -XepExcludedPaths:.*/generated-sources/.*</arg>
<arg>-J--add-exports=jdk.compiler/com.sun.tools.javac.api=ALL-UNNAMED</arg> <arg>-J--add-exports=jdk.compiler/com.sun.tools.javac.api=ALL-UNNAMED</arg>
<arg>-J--add-exports=jdk.compiler/com.sun.tools.javac.file=ALL-UNNAMED</arg> <arg>-J--add-exports=jdk.compiler/com.sun.tools.javac.file=ALL-UNNAMED</arg>
<arg>-J--add-exports=jdk.compiler/com.sun.tools.javac.main=ALL-UNNAMED</arg> <arg>-J--add-exports=jdk.compiler/com.sun.tools.javac.main=ALL-UNNAMED</arg>

View File

@ -198,7 +198,7 @@ public class RaceOnCursorIteratorTest extends ActiveMQTestBase {
final RoutingContextImpl ctx = new RoutingContextImpl(null); final RoutingContextImpl ctx = new RoutingContextImpl(null);
ctx.addQueue(ADDRESS, queue); ctx.addQueue(ADDRESS, queue);
pagingStore.page(msg, ctx.getTransaction(), ctx.getContextListing(ADDRESS), lock); pagingStore.page(msg, ctx.getTransaction(), ctx.getContextListing(ADDRESS));
return msg; return msg;
} }

View File

@ -23,7 +23,6 @@ import java.util.concurrent.CountDownLatch;
import java.util.concurrent.LinkedBlockingDeque; import java.util.concurrent.LinkedBlockingDeque;
import java.util.concurrent.TimeUnit; import java.util.concurrent.TimeUnit;
import java.util.concurrent.atomic.AtomicInteger; import java.util.concurrent.atomic.AtomicInteger;
import java.util.concurrent.locks.ReentrantReadWriteLock;
import org.apache.activemq.artemis.api.config.ActiveMQDefaultConfiguration; import org.apache.activemq.artemis.api.config.ActiveMQDefaultConfiguration;
import org.apache.activemq.artemis.api.core.Message; import org.apache.activemq.artemis.api.core.Message;
@ -354,8 +353,7 @@ public class PersistMultiThreadTest extends ActiveMQTestBase {
@Override @Override
public boolean page(Message message, public boolean page(Message message,
Transaction tx, Transaction tx,
RouteContextList listCtx, RouteContextList listCtx) throws Exception {
ReentrantReadWriteLock.ReadLock readLock) throws Exception {
return false; return false;
} }

View File

@ -408,7 +408,7 @@ public class PageCursorStressTest extends ActiveMQTestBase {
msg.getBodyBuffer().writeBytes(buffer, 0, buffer.writerIndex()); msg.getBodyBuffer().writeBytes(buffer, 0, buffer.writerIndex());
Assert.assertTrue(pageStore.page(msg, ctx.getTransaction(), ctx.getContextListing(ADDRESS), lock)); Assert.assertTrue(pageStore.page(msg, ctx.getTransaction(), ctx.getContextListing(ADDRESS)));
PagedReference readMessage = iterator.next(); PagedReference readMessage = iterator.next();
@ -441,7 +441,7 @@ public class PageCursorStressTest extends ActiveMQTestBase {
msg.getBodyBuffer().writeBytes(buffer, 0, buffer.writerIndex()); msg.getBodyBuffer().writeBytes(buffer, 0, buffer.writerIndex());
Assert.assertTrue(pageStore.page(msg, ctx.getTransaction(), ctx.getContextListing(ADDRESS), lock)); Assert.assertTrue(pageStore.page(msg, ctx.getTransaction(), ctx.getContextListing(ADDRESS)));
} }
PagedReference readMessage = iterator.next(); PagedReference readMessage = iterator.next();
@ -471,7 +471,7 @@ public class PageCursorStressTest extends ActiveMQTestBase {
msg.getBodyBuffer().writeBytes(buffer, 0, buffer.writerIndex()); msg.getBodyBuffer().writeBytes(buffer, 0, buffer.writerIndex());
Assert.assertTrue(pageStore.page(msg, ctx.getTransaction(), ctx.getContextListing(ADDRESS), lock)); Assert.assertTrue(pageStore.page(msg, ctx.getTransaction(), ctx.getContextListing(ADDRESS)));
} }
PagedReference readMessage = iterator.next(); PagedReference readMessage = iterator.next();
@ -556,7 +556,7 @@ public class PageCursorStressTest extends ActiveMQTestBase {
msg.getBodyBuffer().writeBytes(buffer, 0, buffer.writerIndex()); msg.getBodyBuffer().writeBytes(buffer, 0, buffer.writerIndex());
Assert.assertTrue(pageStore.page(msg, ctx.getTransaction(), ctx.getContextListing(ADDRESS), lock)); Assert.assertTrue(pageStore.page(msg, ctx.getTransaction(), ctx.getContextListing(ADDRESS)));
} }
if (tx != null) { if (tx != null) {
@ -775,7 +775,7 @@ public class PageCursorStressTest extends ActiveMQTestBase {
msg.getBodyBuffer().writeBytes(buffer, 0, buffer.writerIndex()); msg.getBodyBuffer().writeBytes(buffer, 0, buffer.writerIndex());
Assert.assertTrue(pageStore.page(msg, ctx.getTransaction(), ctx.getContextListing(ADDRESS), lock)); Assert.assertTrue(pageStore.page(msg, ctx.getTransaction(), ctx.getContextListing(ADDRESS)));
} }
return pageStore.getNumberOfPages(); return pageStore.getNumberOfPages();
@ -872,7 +872,7 @@ public class PageCursorStressTest extends ActiveMQTestBase {
Message msg = new CoreMessage(storage.generateID(), buffer.writerIndex()); Message msg = new CoreMessage(storage.generateID(), buffer.writerIndex());
msg.getBodyBuffer().writeBytes(buffer, 0, buffer.writerIndex()); msg.getBodyBuffer().writeBytes(buffer, 0, buffer.writerIndex());
msg.putIntProperty("key", i); msg.putIntProperty("key", i);
pageStore.page(msg, ctx.getTransaction(), ctx.getContextListing(ADDRESS), lock); pageStore.page(msg, ctx.getTransaction(), ctx.getContextListing(ADDRESS));
} }
return txImpl; return txImpl;

View File

@ -66,11 +66,11 @@ public class PagingManagerImplTest extends ActiveMQTestBase {
ICoreMessage msg = createMessage(1L, new SimpleString("simple-test"), createRandomBuffer(10)); ICoreMessage msg = createMessage(1L, new SimpleString("simple-test"), createRandomBuffer(10));
final RoutingContextImpl ctx = new RoutingContextImpl(null); final RoutingContextImpl ctx = new RoutingContextImpl(null);
Assert.assertFalse(store.page(msg, ctx.getTransaction(), ctx.getContextListing(store.getStoreName()), lock)); Assert.assertFalse(store.page(msg, ctx.getTransaction(), ctx.getContextListing(store.getStoreName())));
store.startPaging(); store.startPaging();
Assert.assertTrue(store.page(msg, ctx.getTransaction(), ctx.getContextListing(store.getStoreName()), lock)); Assert.assertTrue(store.page(msg, ctx.getTransaction(), ctx.getContextListing(store.getStoreName())));
Page page = store.depage(); Page page = store.depage();
@ -89,7 +89,7 @@ public class PagingManagerImplTest extends ActiveMQTestBase {
Assert.assertNull(store.depage()); Assert.assertNull(store.depage());
final RoutingContextImpl ctx2 = new RoutingContextImpl(null); final RoutingContextImpl ctx2 = new RoutingContextImpl(null);
Assert.assertFalse(store.page(msg, ctx2.getTransaction(), ctx2.getContextListing(store.getStoreName()), lock)); Assert.assertFalse(store.page(msg, ctx2.getTransaction(), ctx2.getContextListing(store.getStoreName())));
} }

View File

@ -176,7 +176,7 @@ public class PagingStoreImplTest extends ActiveMQTestBase {
Assert.assertTrue(storeImpl.isPaging()); Assert.assertTrue(storeImpl.isPaging());
final RoutingContextImpl ctx = new RoutingContextImpl(null); final RoutingContextImpl ctx = new RoutingContextImpl(null);
Assert.assertTrue(storeImpl.page(msg, ctx.getTransaction(), ctx.getContextListing(storeImpl.getStoreName()), lock)); Assert.assertTrue(storeImpl.page(msg, ctx.getTransaction(), ctx.getContextListing(storeImpl.getStoreName())));
Assert.assertEquals(1, storeImpl.getNumberOfPages()); Assert.assertEquals(1, storeImpl.getNumberOfPages());
@ -219,7 +219,7 @@ public class PagingStoreImplTest extends ActiveMQTestBase {
Message msg = createMessage(i, storeImpl, destination, buffer); Message msg = createMessage(i, storeImpl, destination, buffer);
final RoutingContextImpl ctx = new RoutingContextImpl(null); final RoutingContextImpl ctx = new RoutingContextImpl(null);
Assert.assertTrue(storeImpl.page(msg, ctx.getTransaction(), ctx.getContextListing(storeImpl.getStoreName()), lock)); Assert.assertTrue(storeImpl.page(msg, ctx.getTransaction(), ctx.getContextListing(storeImpl.getStoreName())));
} }
@ -286,7 +286,7 @@ public class PagingStoreImplTest extends ActiveMQTestBase {
msg.putIntProperty("page", page); msg.putIntProperty("page", page);
final RoutingContextImpl ctx = new RoutingContextImpl(null); final RoutingContextImpl ctx = new RoutingContextImpl(null);
ctx.addQueue(fakeQueue.getName(), fakeQueue); ctx.addQueue(fakeQueue.getName(), fakeQueue);
Assert.assertTrue(storeImpl.page(msg, ctx.getTransaction(), ctx.getContextListing(storeImpl.getStoreName()), lock)); Assert.assertTrue(storeImpl.page(msg, ctx.getTransaction(), ctx.getContextListing(storeImpl.getStoreName())));
if (i > 0 && i % 10 == 0) { if (i > 0 && i % 10 == 0) {
storeImpl.forceAnotherPage(); storeImpl.forceAnotherPage();
page++; page++;
@ -372,7 +372,7 @@ public class PagingStoreImplTest extends ActiveMQTestBase {
msg.putIntProperty("page", page); msg.putIntProperty("page", page);
final RoutingContextImpl ctx = new RoutingContextImpl(null); final RoutingContextImpl ctx = new RoutingContextImpl(null);
ctx.addQueue(fakeQueue.getName(), fakeQueue); ctx.addQueue(fakeQueue.getName(), fakeQueue);
Assert.assertTrue(storeImpl.page(msg, ctx.getTransaction(), ctx.getContextListing(storeImpl.getStoreName()), lock)); Assert.assertTrue(storeImpl.page(msg, ctx.getTransaction(), ctx.getContextListing(storeImpl.getStoreName())));
if (i > 0 && i % 10 == 0) { if (i > 0 && i % 10 == 0) {
storeImpl.forceAnotherPage(); storeImpl.forceAnotherPage();
page++; page++;
@ -474,7 +474,7 @@ public class PagingStoreImplTest extends ActiveMQTestBase {
Message msg = createMessage(i, store, destination, buffer); Message msg = createMessage(i, store, destination, buffer);
final RoutingContextImpl ctx = new RoutingContextImpl(null); final RoutingContextImpl ctx = new RoutingContextImpl(null);
Assert.assertTrue(store.page(msg, ctx.getTransaction(), ctx.getContextListing(store.getStoreName()), lock)); Assert.assertTrue(store.page(msg, ctx.getTransaction(), ctx.getContextListing(store.getStoreName())));
} }
Assert.assertEquals(2, store.getNumberOfPages()); Assert.assertEquals(2, store.getNumberOfPages());
@ -509,7 +509,7 @@ public class PagingStoreImplTest extends ActiveMQTestBase {
Message msg = createMessage(1, store, destination, buffers.get(0)); Message msg = createMessage(1, store, destination, buffers.get(0));
final RoutingContextImpl ctx = new RoutingContextImpl(null); final RoutingContextImpl ctx = new RoutingContextImpl(null);
Assert.assertTrue(store.page(msg, ctx.getTransaction(), ctx.getContextListing(store.getStoreName()), lock)); Assert.assertTrue(store.page(msg, ctx.getTransaction(), ctx.getContextListing(store.getStoreName())));
Page newPage = store.depage(); Page newPage = store.depage();
@ -529,14 +529,14 @@ public class PagingStoreImplTest extends ActiveMQTestBase {
{ {
final RoutingContextImpl ctx2 = new RoutingContextImpl(null); final RoutingContextImpl ctx2 = new RoutingContextImpl(null);
Assert.assertFalse(store.page(msg, ctx2.getTransaction(), ctx2.getContextListing(store.getStoreName()), lock)); Assert.assertFalse(store.page(msg, ctx2.getTransaction(), ctx2.getContextListing(store.getStoreName())));
} }
store.startPaging(); store.startPaging();
{ {
final RoutingContextImpl ctx2 = new RoutingContextImpl(null); final RoutingContextImpl ctx2 = new RoutingContextImpl(null);
Assert.assertTrue(store.page(msg, ctx2.getTransaction(), ctx2.getContextListing(store.getStoreName()), lock)); Assert.assertTrue(store.page(msg, ctx2.getTransaction(), ctx2.getContextListing(store.getStoreName())));
} }
Page page = store.depage(); Page page = store.depage();
@ -619,7 +619,7 @@ public class PagingStoreImplTest extends ActiveMQTestBase {
// Just using the internal API to remove it from the page file system // Just using the internal API to remove it from the page file system
Message msg = createMessage(id, storeImpl, destination, createRandomBuffer(id, 5)); Message msg = createMessage(id, storeImpl, destination, createRandomBuffer(id, 5));
final RoutingContextImpl ctx2 = new RoutingContextImpl(null); final RoutingContextImpl ctx2 = new RoutingContextImpl(null);
if (storeImpl.page(msg, ctx2.getTransaction(), ctx2.getContextListing(storeImpl.getStoreName()), lock)) { if (storeImpl.page(msg, ctx2.getTransaction(), ctx2.getContextListing(storeImpl.getStoreName()))) {
buffers.put(id, msg); buffers.put(id, msg);
} else { } else {
break; break;
@ -736,7 +736,7 @@ public class PagingStoreImplTest extends ActiveMQTestBase {
storeImpl2.forceAnotherPage(); storeImpl2.forceAnotherPage();
final RoutingContextImpl ctx = new RoutingContextImpl(null); final RoutingContextImpl ctx = new RoutingContextImpl(null);
storeImpl2.page(lastMsg, ctx.getTransaction(), ctx.getContextListing(storeImpl2.getStoreName()), lock); storeImpl2.page(lastMsg, ctx.getTransaction(), ctx.getContextListing(storeImpl2.getStoreName()));
buffers2.put(lastMessageId, lastMsg); buffers2.put(lastMessageId, lastMsg);
Page lastPage = null; Page lastPage = null;
@ -858,7 +858,7 @@ public class PagingStoreImplTest extends ActiveMQTestBase {
msg.putLongProperty("count", i); msg.putLongProperty("count", i);
final RoutingContextImpl ctx2 = new RoutingContextImpl(null); final RoutingContextImpl ctx2 = new RoutingContextImpl(null);
while (!store.page(msg, ctx2.getTransaction(), ctx2.getContextListing(store.getStoreName()), lock)) { while (!store.page(msg, ctx2.getTransaction(), ctx2.getContextListing(store.getStoreName()))) {
store.startPaging(); store.startPaging();
} }
@ -1282,7 +1282,7 @@ public class PagingStoreImplTest extends ActiveMQTestBase {
msg.putLongProperty("count", id); msg.putLongProperty("count", id);
final RoutingContextImpl ctx2 = new RoutingContextImpl(null); final RoutingContextImpl ctx2 = new RoutingContextImpl(null);
storeImpl.page(msg, ctx2.getTransaction(), ctx2.getContextListing(storeImpl.getStoreName()), lock); storeImpl.page(msg, ctx2.getTransaction(), ctx2.getContextListing(storeImpl.getStoreName()));
} }
private CoreMessage createMessage(final long id, private CoreMessage createMessage(final long id,