#7382 Signal Queue not empty condition on every append as is standard BlockingQueue behaviour

Fixes #7380
This commit is contained in:
Armin 2017-06-08 10:18:34 +02:00 committed by Armin Braun
parent b6e051acbc
commit 87223c6701
2 changed files with 51 additions and 6 deletions

View file

@ -324,7 +324,6 @@ public class Queue implements Closeable {
lock.lock();
try {
boolean wasEmpty = (firstUnreadPage() == null);
// create a new head page if the current does not have sufficient space left for data to be written
if (! this.headPage.hasSpace(data.length)) {
@ -356,11 +355,8 @@ public class Queue implements Closeable {
long seqNum = nextSeqNum();
this.headPage.write(data, seqNum, this.checkpointMaxWrites);
this.unreadCount++;
// if the queue was empty before write, signal non emptiness
// a simple signal and not signalAll is necessary here since writing a single element
// can only really enable a single thread to read a batch
if (wasEmpty) { notEmpty.signal(); }
notEmpty.signal();
// now check if we reached a queue full state and block here until it is not full
// for the next write or the queue was closed.

View file

@ -13,6 +13,7 @@ import java.util.concurrent.ExecutorService;
import java.util.concurrent.Executors;
import java.util.concurrent.Future;
import java.util.concurrent.TimeUnit;
import java.util.concurrent.TimeoutException;
import java.util.concurrent.atomic.AtomicInteger;
import org.junit.After;
import org.junit.Before;
@ -561,6 +562,54 @@ public class QueueTest {
}
}
@Test
public void queueStableUnderStress() throws Exception {
Settings settings = TestSettings.persistedQueueSettings(1000000, dataPath);
final ExecutorService exec = Executors.newScheduledThreadPool(2);
try (Queue queue = new Queue(settings)) {
final int count = 20_000;
final int concurrent = 2;
queue.open();
final Future<Integer>[] futures = new Future[concurrent];
for (int c = 0; c < concurrent; ++c) {
futures[c] = exec.submit(() -> {
int i = 0;
try {
while (i < count / concurrent) {
final Batch batch = queue.readBatch(1);
for (final Queueable elem : batch.getElements()) {
if (elem != null) {
++i;
}
}
}
return i;
} catch (final IOException ex) {
throw new IllegalStateException(ex);
}
});
}
for (int i = 0; i < count; ++i) {
try {
final Queueable evnt = new StringElement("foo");
queue.write(evnt);
} catch (final IOException ex) {
throw new IllegalStateException(ex);
}
}
assertThat(
Arrays.stream(futures).map(i -> {
try {
return i.get(10L, TimeUnit.SECONDS);
} catch (final InterruptedException | ExecutionException | TimeoutException ex) {
throw new IllegalStateException(ex);
}
}).reduce((x, y) -> x + y).orElse(0),
is(20_000)
);
}
}
@Test
public void testAckedCount() throws IOException {
Settings settings = TestSettings.persistedQueueSettings(100, dataPath);