This closes #2270
This commit is contained in:
commit
f8140b91d4
|
@ -241,7 +241,7 @@ public abstract class AbstractJournalUpdateTask implements JournalReaderCallback
|
||||||
|
|
||||||
writingChannel = ActiveMQBuffers.wrappedBuffer(bufferWrite);
|
writingChannel = ActiveMQBuffers.wrappedBuffer(bufferWrite);
|
||||||
|
|
||||||
currentFile = filesRepository.takeFile(false, false, false, true);
|
currentFile = filesRepository.openFileCMP();
|
||||||
|
|
||||||
sequentialFile = currentFile.getFile();
|
sequentialFile = currentFile.getFile();
|
||||||
|
|
||||||
|
|
|
@ -409,6 +409,16 @@ public class JournalFilesRepository {
|
||||||
return openedFiles.size();
|
return openedFiles.size();
|
||||||
}
|
}
|
||||||
|
|
||||||
|
public JournalFile openFileCMP() throws Exception {
|
||||||
|
JournalFile file = openFile();
|
||||||
|
|
||||||
|
SequentialFile sequentialFile = file.getFile();
|
||||||
|
sequentialFile.close();
|
||||||
|
sequentialFile.renameTo(sequentialFile.getFileName() + ".cmp");
|
||||||
|
|
||||||
|
return file;
|
||||||
|
}
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* <p>This method will instantly return the opened file, and schedule opening and reclaiming.</p>
|
* <p>This method will instantly return the opened file, and schedule opening and reclaiming.</p>
|
||||||
* <p>In case there are no cached opened files, this method will block until the file was opened,
|
* <p>In case there are no cached opened files, this method will block until the file was opened,
|
||||||
|
@ -468,7 +478,7 @@ public class JournalFilesRepository {
|
||||||
/**
|
/**
|
||||||
* Open a file and place it into the openedFiles queue
|
* Open a file and place it into the openedFiles queue
|
||||||
*/
|
*/
|
||||||
public void pushOpenedFile() throws Exception {
|
public synchronized void pushOpenedFile() throws Exception {
|
||||||
JournalFile nextOpenedFile = takeFile(true, true, true, false);
|
JournalFile nextOpenedFile = takeFile(true, true, true, false);
|
||||||
|
|
||||||
if (logger.isTraceEnabled()) {
|
if (logger.isTraceEnabled()) {
|
||||||
|
@ -505,7 +515,7 @@ public class JournalFilesRepository {
|
||||||
* @throws Exception
|
* @throws Exception
|
||||||
* @see JournalImpl#initFileHeader(SequentialFileFactory, SequentialFile, int, long)
|
* @see JournalImpl#initFileHeader(SequentialFileFactory, SequentialFile, int, long)
|
||||||
*/
|
*/
|
||||||
public JournalFile takeFile(final boolean keepOpened,
|
private JournalFile takeFile(final boolean keepOpened,
|
||||||
final boolean multiAIO,
|
final boolean multiAIO,
|
||||||
final boolean initFile,
|
final boolean initFile,
|
||||||
final boolean tmpCompactExtension) throws Exception {
|
final boolean tmpCompactExtension) throws Exception {
|
||||||
|
|
|
@ -348,7 +348,12 @@ public class JournalImpl extends JournalBase implements TestableJournal, Journal
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
public String toString() {
|
public String toString() {
|
||||||
return "JournalImpl(state=" + state + ", currentFile=[" + currentFile + "], hash=" + super.toString() + ")";
|
try {
|
||||||
|
return "JournalImpl(state=" + state + ", directory=[" + this.fileFactory.getDirectory().toString() + "], hash=" + super.toString() + ")";
|
||||||
|
} catch (Throwable e) {
|
||||||
|
logger.warn(e);
|
||||||
|
return super.toString();
|
||||||
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
|
@ -1278,6 +1283,9 @@ public class JournalImpl extends JournalBase implements TestableJournal, Journal
|
||||||
JournalInternalRecord commitRecord = new JournalCompleteRecordTX(TX_RECORD_TYPE.COMMIT, txID, null);
|
JournalInternalRecord commitRecord = new JournalCompleteRecordTX(TX_RECORD_TYPE.COMMIT, txID, null);
|
||||||
JournalFile usedFile = appendRecord(commitRecord, true, sync, tx, callback);
|
JournalFile usedFile = appendRecord(commitRecord, true, sync, tx, callback);
|
||||||
|
|
||||||
|
if (logger.isTraceEnabled()) {
|
||||||
|
logger.trace("appendCommitRecord::txID=" + txID + ", usedFile = " + usedFile);
|
||||||
|
}
|
||||||
|
|
||||||
tx.commit(usedFile);
|
tx.commit(usedFile);
|
||||||
} catch (ActiveMQShutdownException e) {
|
} catch (ActiveMQShutdownException e) {
|
||||||
|
@ -1417,7 +1425,7 @@ public class JournalImpl extends JournalBase implements TestableJournal, Journal
|
||||||
private void checkDeleteSize() {
|
private void checkDeleteSize() {
|
||||||
// HORNETQ-482 - Flush deletes only if memory is critical
|
// HORNETQ-482 - Flush deletes only if memory is critical
|
||||||
if (recordsToDelete.size() > DELETE_FLUSH && runtime.freeMemory() < runtime.maxMemory() * 0.2) {
|
if (recordsToDelete.size() > DELETE_FLUSH && runtime.freeMemory() < runtime.maxMemory() * 0.2) {
|
||||||
ActiveMQJournalLogger.LOGGER.debug("Flushing deletes during loading, deleteCount = " + recordsToDelete.size());
|
logger.debug("Flushing deletes during loading, deleteCount = " + recordsToDelete.size());
|
||||||
// Clean up when the list is too large, or it won't be possible to load large sets of files
|
// Clean up when the list is too large, or it won't be possible to load large sets of files
|
||||||
// Done as part of JBMESSAGING-1678
|
// Done as part of JBMESSAGING-1678
|
||||||
Iterator<RecordInfo> iter = records.iterator();
|
Iterator<RecordInfo> iter = records.iterator();
|
||||||
|
@ -1431,7 +1439,7 @@ public class JournalImpl extends JournalBase implements TestableJournal, Journal
|
||||||
|
|
||||||
recordsToDelete.clear();
|
recordsToDelete.clear();
|
||||||
|
|
||||||
ActiveMQJournalLogger.LOGGER.debug("flush delete done");
|
logger.debug("flush delete done");
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
|
@ -1529,8 +1537,8 @@ public class JournalImpl extends JournalBase implements TestableJournal, Journal
|
||||||
throw new IllegalStateException("There is pending compacting operation");
|
throw new IllegalStateException("There is pending compacting operation");
|
||||||
}
|
}
|
||||||
|
|
||||||
if (ActiveMQJournalLogger.LOGGER.isDebugEnabled()) {
|
if (logger.isDebugEnabled()) {
|
||||||
ActiveMQJournalLogger.LOGGER.debug("JournalImpl::compact compacting journal " + (++compactCount));
|
logger.debug("JournalImpl::compact " + JournalImpl.this + " for its " + (++compactCount) + " time");
|
||||||
}
|
}
|
||||||
|
|
||||||
compactorLock.writeLock().lock();
|
compactorLock.writeLock().lock();
|
||||||
|
@ -1540,7 +1548,9 @@ public class JournalImpl extends JournalBase implements TestableJournal, Journal
|
||||||
boolean previousReclaimValue = isAutoReclaim();
|
boolean previousReclaimValue = isAutoReclaim();
|
||||||
|
|
||||||
try {
|
try {
|
||||||
ActiveMQJournalLogger.LOGGER.debug("Starting compacting operation on journal");
|
if (logger.isDebugEnabled()) {
|
||||||
|
logger.debug("Starting compacting operation on journal " + this);
|
||||||
|
}
|
||||||
|
|
||||||
onCompactStart();
|
onCompactStart();
|
||||||
|
|
||||||
|
@ -1669,9 +1679,14 @@ public class JournalImpl extends JournalBase implements TestableJournal, Journal
|
||||||
renameFiles(dataFilesToProcess, newDatafiles);
|
renameFiles(dataFilesToProcess, newDatafiles);
|
||||||
deleteControlFile(controlFile);
|
deleteControlFile(controlFile);
|
||||||
|
|
||||||
ActiveMQJournalLogger.LOGGER.debug("Finished compacting on journal");
|
if (logger.isDebugEnabled()) {
|
||||||
|
logger.debug("Finished compacting on journal " + this);
|
||||||
|
}
|
||||||
|
|
||||||
} finally {
|
} finally {
|
||||||
|
if (logger.isDebugEnabled()) {
|
||||||
|
logger.debug("Flushing compacting on journal " + this);
|
||||||
|
}
|
||||||
// An Exception was probably thrown, and the compactor was not cleared
|
// An Exception was probably thrown, and the compactor was not cleared
|
||||||
if (compactor != null) {
|
if (compactor != null) {
|
||||||
try {
|
try {
|
||||||
|
@ -1681,12 +1696,15 @@ public class JournalImpl extends JournalBase implements TestableJournal, Journal
|
||||||
|
|
||||||
compactor = null;
|
compactor = null;
|
||||||
}
|
}
|
||||||
|
if (logger.isDebugEnabled()) {
|
||||||
|
logger.debug("since compact finished, setAutoReclaim back into " + previousReclaimValue);
|
||||||
|
}
|
||||||
setAutoReclaim(previousReclaimValue);
|
setAutoReclaim(previousReclaimValue);
|
||||||
}
|
}
|
||||||
} finally {
|
} finally {
|
||||||
compactorLock.writeLock().unlock();
|
compactorLock.writeLock().unlock();
|
||||||
if (ActiveMQJournalLogger.LOGGER.isDebugEnabled()) {
|
if (logger.isDebugEnabled()) {
|
||||||
ActiveMQJournalLogger.LOGGER.debug("JournalImpl::compact finishing");
|
logger.debug("JournalImpl::compact finalized");
|
||||||
}
|
}
|
||||||
|
|
||||||
|
|
||||||
|
|
|
@ -238,13 +238,13 @@ public class JournalTransaction {
|
||||||
// without setting this properly...
|
// without setting this properly...
|
||||||
if (compacting && compactor != null) {
|
if (compacting && compactor != null) {
|
||||||
if (logger.isTraceEnabled()) {
|
if (logger.isTraceEnabled()) {
|
||||||
logger.trace("adding tx " + this.id + " into compacting");
|
logger.trace("adding txID=" + this.id + " into compacting");
|
||||||
}
|
}
|
||||||
compactor.addCommandCommit(this, file);
|
compactor.addCommandCommit(this, file);
|
||||||
} else {
|
} else {
|
||||||
|
|
||||||
if (logger.isTraceEnabled()) {
|
if (logger.isTraceEnabled()) {
|
||||||
logger.trace("no compact commit " + this.id);
|
logger.trace("there was no compactor on commit txID=" + this.id);
|
||||||
}
|
}
|
||||||
if (pos != null) {
|
if (pos != null) {
|
||||||
for (JournalUpdate trUpdate : pos) {
|
for (JournalUpdate trUpdate : pos) {
|
||||||
|
|
|
@ -669,6 +669,9 @@ public abstract class AbstractJournalStorageManager extends CriticalComponentImp
|
||||||
try {
|
try {
|
||||||
messageJournal.appendCommitRecord(txID, syncTransactional, getContext(syncTransactional), lineUpContext);
|
messageJournal.appendCommitRecord(txID, syncTransactional, getContext(syncTransactional), lineUpContext);
|
||||||
if (!lineUpContext && !syncTransactional) {
|
if (!lineUpContext && !syncTransactional) {
|
||||||
|
if (logger.isTraceEnabled()) {
|
||||||
|
logger.trace("calling getContext(true).done() for txID=" + txID + ",lineupContext=" + lineUpContext + " syncTransactional=" + syncTransactional + "... forcing call on getContext(true).done");
|
||||||
|
}
|
||||||
/**
|
/**
|
||||||
* If {@code lineUpContext == false}, it means that we have previously lined up a
|
* If {@code lineUpContext == false}, it means that we have previously lined up a
|
||||||
* context somewhere else (specifically see @{link TransactionImpl#asyncAppendCommit}),
|
* context somewhere else (specifically see @{link TransactionImpl#asyncAppendCommit}),
|
||||||
|
@ -1742,7 +1745,9 @@ public abstract class AbstractJournalStorageManager extends CriticalComponentImp
|
||||||
|
|
||||||
if (record.isUpdate) {
|
if (record.isUpdate) {
|
||||||
PageTransactionInfo pgTX = pagingManager.getTransaction(pageTransactionInfo.getTransactionID());
|
PageTransactionInfo pgTX = pagingManager.getTransaction(pageTransactionInfo.getTransactionID());
|
||||||
pgTX.reloadUpdate(this, pagingManager, tx, pageTransactionInfo.getNumberOfMessages());
|
if (pgTX != null) {
|
||||||
|
pgTX.reloadUpdate(this, pagingManager, tx, pageTransactionInfo.getNumberOfMessages());
|
||||||
|
}
|
||||||
} else {
|
} else {
|
||||||
pageTransactionInfo.setCommitted(false);
|
pageTransactionInfo.setCommitted(false);
|
||||||
|
|
||||||
|
|
|
@ -149,7 +149,7 @@ public class ReplicatedJournal implements Journal {
|
||||||
final Persister persister,
|
final Persister persister,
|
||||||
final Object record) throws Exception {
|
final Object record) throws Exception {
|
||||||
if (log.isTraceEnabled()) {
|
if (log.isTraceEnabled()) {
|
||||||
log.trace("Append record TXid = " + id + " recordType = " + recordType);
|
log.trace("Append record txID=" + id + " recordType = " + recordType);
|
||||||
}
|
}
|
||||||
replicationManager.appendAddRecordTransactional(journalID, ADD_OPERATION_TYPE.ADD, txID, id, recordType, persister, record);
|
replicationManager.appendAddRecordTransactional(journalID, ADD_OPERATION_TYPE.ADD, txID, id, recordType, persister, record);
|
||||||
localJournal.appendAddRecordTransactional(txID, id, recordType, persister, record);
|
localJournal.appendAddRecordTransactional(txID, id, recordType, persister, record);
|
||||||
|
@ -164,7 +164,7 @@ public class ReplicatedJournal implements Journal {
|
||||||
@Override
|
@Override
|
||||||
public void appendCommitRecord(final long txID, final boolean sync) throws Exception {
|
public void appendCommitRecord(final long txID, final boolean sync) throws Exception {
|
||||||
if (log.isTraceEnabled()) {
|
if (log.isTraceEnabled()) {
|
||||||
log.trace("AppendCommit " + txID);
|
log.trace("AppendCommit txID=" + txID);
|
||||||
}
|
}
|
||||||
replicationManager.appendCommitRecord(journalID, txID, sync, true);
|
replicationManager.appendCommitRecord(journalID, txID, sync, true);
|
||||||
localJournal.appendCommitRecord(txID, sync);
|
localJournal.appendCommitRecord(txID, sync);
|
||||||
|
@ -516,8 +516,8 @@ public class ReplicatedJournal implements Journal {
|
||||||
}
|
}
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
public void forceMoveNextFile() {
|
public void forceMoveNextFile() throws Exception {
|
||||||
throw new UnsupportedOperationException();
|
localJournal.forceMoveNextFile();
|
||||||
}
|
}
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
|
|
|
@ -27,7 +27,6 @@ import java.util.Set;
|
||||||
import java.util.concurrent.ConcurrentLinkedQueue;
|
import java.util.concurrent.ConcurrentLinkedQueue;
|
||||||
import java.util.concurrent.Executor;
|
import java.util.concurrent.Executor;
|
||||||
import java.util.concurrent.TimeUnit;
|
import java.util.concurrent.TimeUnit;
|
||||||
import java.util.concurrent.atomic.AtomicBoolean;
|
|
||||||
|
|
||||||
import io.netty.buffer.ByteBuf;
|
import io.netty.buffer.ByteBuf;
|
||||||
import io.netty.buffer.PooledByteBufAllocator;
|
import io.netty.buffer.PooledByteBufAllocator;
|
||||||
|
@ -116,8 +115,6 @@ public final class ReplicationManager implements ActiveMQComponent {
|
||||||
|
|
||||||
private volatile boolean enabled;
|
private volatile boolean enabled;
|
||||||
|
|
||||||
private final AtomicBoolean writable = new AtomicBoolean(true);
|
|
||||||
|
|
||||||
private final Queue<OperationContext> pendingTokens = new ConcurrentLinkedQueue<>();
|
private final Queue<OperationContext> pendingTokens = new ConcurrentLinkedQueue<>();
|
||||||
|
|
||||||
private final ExecutorFactory ioExecutorFactory;
|
private final ExecutorFactory ioExecutorFactory;
|
||||||
|
@ -291,6 +288,12 @@ public final class ReplicationManager implements ActiveMQComponent {
|
||||||
logger.trace("Stopping being ignored as it hasn't been started");
|
logger.trace("Stopping being ignored as it hasn't been started");
|
||||||
return;
|
return;
|
||||||
}
|
}
|
||||||
|
|
||||||
|
started = false;
|
||||||
|
}
|
||||||
|
|
||||||
|
if (logger.isTraceEnabled()) {
|
||||||
|
logger.trace("stop(clearTokens=" + clearTokens + ")", new Exception("Trace"));
|
||||||
}
|
}
|
||||||
|
|
||||||
// This is to avoid the write holding a lock while we are trying to close it
|
// This is to avoid the write holding a lock while we are trying to close it
|
||||||
|
@ -300,7 +303,6 @@ public final class ReplicationManager implements ActiveMQComponent {
|
||||||
}
|
}
|
||||||
|
|
||||||
enabled = false;
|
enabled = false;
|
||||||
writable.set(true);
|
|
||||||
|
|
||||||
if (clearTokens) {
|
if (clearTokens) {
|
||||||
clearReplicationTokens();
|
clearReplicationTokens();
|
||||||
|
@ -312,7 +314,6 @@ public final class ReplicationManager implements ActiveMQComponent {
|
||||||
toStop.destroy();
|
toStop.destroy();
|
||||||
}
|
}
|
||||||
remotingConnection = null;
|
remotingConnection = null;
|
||||||
started = false;
|
|
||||||
}
|
}
|
||||||
|
|
||||||
/**
|
/**
|
||||||
|
|
|
@ -976,6 +976,8 @@ public class ActiveMQServerImpl implements ActiveMQServer {
|
||||||
*/
|
*/
|
||||||
void stop(boolean failoverOnServerShutdown, final boolean criticalIOError, boolean restarting, boolean isShutdown) {
|
void stop(boolean failoverOnServerShutdown, final boolean criticalIOError, boolean restarting, boolean isShutdown) {
|
||||||
|
|
||||||
|
logger.debug("Stopping server");
|
||||||
|
|
||||||
synchronized (this) {
|
synchronized (this) {
|
||||||
if (state == SERVER_STATE.STOPPED || state == SERVER_STATE.STOPPING) {
|
if (state == SERVER_STATE.STOPPED || state == SERVER_STATE.STOPPING) {
|
||||||
return;
|
return;
|
||||||
|
|
|
@ -616,7 +616,7 @@ public class TransactionImpl implements Transaction {
|
||||||
public String toString() {
|
public String toString() {
|
||||||
Date dt = new Date(this.createTime);
|
Date dt = new Date(this.createTime);
|
||||||
return "TransactionImpl [xid=" + xid +
|
return "TransactionImpl [xid=" + xid +
|
||||||
", id=" +
|
", txID=" +
|
||||||
id +
|
id +
|
||||||
", xid=" + xid +
|
", xid=" + xid +
|
||||||
", state=" +
|
", state=" +
|
||||||
|
|
|
@ -0,0 +1,93 @@
|
||||||
|
/*
|
||||||
|
* 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.unit.core.journal.impl;
|
||||||
|
|
||||||
|
import java.util.LinkedList;
|
||||||
|
import java.util.concurrent.BlockingDeque;
|
||||||
|
import java.util.concurrent.ExecutorService;
|
||||||
|
import java.util.concurrent.Executors;
|
||||||
|
import java.util.concurrent.LinkedBlockingDeque;
|
||||||
|
import java.util.concurrent.atomic.AtomicBoolean;
|
||||||
|
|
||||||
|
import org.apache.activemq.artemis.core.journal.impl.JournalFile;
|
||||||
|
import org.apache.activemq.artemis.core.journal.impl.JournalFilesRepository;
|
||||||
|
import org.apache.activemq.artemis.core.journal.impl.JournalImpl;
|
||||||
|
import org.apache.activemq.artemis.junit.Wait;
|
||||||
|
import org.apache.activemq.artemis.tests.unit.core.journal.impl.fakes.FakeSequentialFileFactory;
|
||||||
|
import org.apache.activemq.artemis.tests.util.ActiveMQTestBase;
|
||||||
|
import org.apache.activemq.artemis.utils.ActiveMQThreadFactory;
|
||||||
|
import org.apache.activemq.artemis.utils.actors.OrderedExecutorFactory;
|
||||||
|
import org.junit.Assert;
|
||||||
|
import org.junit.Test;
|
||||||
|
|
||||||
|
public class JournalFileRepositoryOrderTest extends ActiveMQTestBase {
|
||||||
|
|
||||||
|
@Test
|
||||||
|
public void testOrder() throws Throwable {
|
||||||
|
ExecutorService executorService = Executors.newFixedThreadPool(3, new ActiveMQThreadFactory("test", false, JournalFileRepositoryOrderTest.class.getClassLoader()));
|
||||||
|
final AtomicBoolean running = new AtomicBoolean(true);
|
||||||
|
Thread t = null;
|
||||||
|
try {
|
||||||
|
FakeSequentialFileFactory fakeSequentialFileFactory = new FakeSequentialFileFactory();
|
||||||
|
JournalImpl journal = new JournalImpl(new OrderedExecutorFactory(executorService), 10 * 1024, 2, -1, -1, 0, fakeSequentialFileFactory, "file", "file", 1, 0);
|
||||||
|
|
||||||
|
final JournalFilesRepository repository = journal.getFilesRepository();
|
||||||
|
final BlockingDeque<JournalFile> dataFiles = new LinkedBlockingDeque<>();
|
||||||
|
|
||||||
|
|
||||||
|
// this is simulating how compating would return files into the journal
|
||||||
|
t = new Thread() {
|
||||||
|
@Override
|
||||||
|
public void run() {
|
||||||
|
while (running.get()) {
|
||||||
|
try {
|
||||||
|
Wait.waitFor(() -> !running.get() || dataFiles.size() > 10, 1000, 1);
|
||||||
|
while (running.get()) {
|
||||||
|
JournalFile file = dataFiles.poll();
|
||||||
|
if (file == null) break;
|
||||||
|
repository.addFreeFile(file, false);
|
||||||
|
}
|
||||||
|
} catch (Throwable e) {
|
||||||
|
e.printStackTrace();
|
||||||
|
}
|
||||||
|
}
|
||||||
|
}
|
||||||
|
};
|
||||||
|
t.start();
|
||||||
|
JournalFile file = null;
|
||||||
|
LinkedList<Integer> values = new LinkedList<>();
|
||||||
|
for (int i = 0; i < 5000; i++) {
|
||||||
|
file = repository.openFile();
|
||||||
|
Assert.assertNotNull(file);
|
||||||
|
values.add(file.getRecordID());
|
||||||
|
dataFiles.push(file);
|
||||||
|
}
|
||||||
|
|
||||||
|
int previous = Integer.MIN_VALUE;
|
||||||
|
for (Integer v : values) {
|
||||||
|
Assert.assertTrue(v.intValue() > previous);
|
||||||
|
previous = v;
|
||||||
|
}
|
||||||
|
|
||||||
|
} finally {
|
||||||
|
running.set(false);
|
||||||
|
executorService.shutdownNow();
|
||||||
|
}
|
||||||
|
|
||||||
|
}
|
||||||
|
}
|
Loading…
Reference in New Issue