mirror of
https://github.com/elastic/logstash.git
synced 2025-04-24 14:47:19 -04:00
parent
4926f5b6bb
commit
941446a6dc
1 changed files with 59 additions and 21 deletions
|
@ -10,14 +10,17 @@ import java.util.concurrent.Future;
|
||||||
import java.util.concurrent.TimeUnit;
|
import java.util.concurrent.TimeUnit;
|
||||||
import org.apache.commons.io.FileUtils;
|
import org.apache.commons.io.FileUtils;
|
||||||
import org.logstash.Event;
|
import org.logstash.Event;
|
||||||
import org.logstash.Timestamp;
|
|
||||||
import org.logstash.ackedqueue.Batch;
|
import org.logstash.ackedqueue.Batch;
|
||||||
import org.logstash.ackedqueue.Queue;
|
import org.logstash.ackedqueue.Queue;
|
||||||
import org.logstash.ackedqueue.Queueable;
|
import org.logstash.ackedqueue.Queueable;
|
||||||
import org.logstash.ackedqueue.Settings;
|
import org.logstash.ackedqueue.Settings;
|
||||||
import org.logstash.ackedqueue.SettingsImpl;
|
import org.logstash.ackedqueue.SettingsImpl;
|
||||||
|
import org.logstash.ackedqueue.io.ByteBufferPageIO;
|
||||||
|
import org.logstash.ackedqueue.io.CheckpointIOFactory;
|
||||||
import org.logstash.ackedqueue.io.FileCheckpointIO;
|
import org.logstash.ackedqueue.io.FileCheckpointIO;
|
||||||
|
import org.logstash.ackedqueue.io.MemoryCheckpointIO;
|
||||||
import org.logstash.ackedqueue.io.MmapPageIO;
|
import org.logstash.ackedqueue.io.MmapPageIO;
|
||||||
|
import org.logstash.ackedqueue.io.PageIOFactory;
|
||||||
import org.openjdk.jmh.annotations.Benchmark;
|
import org.openjdk.jmh.annotations.Benchmark;
|
||||||
import org.openjdk.jmh.annotations.BenchmarkMode;
|
import org.openjdk.jmh.annotations.BenchmarkMode;
|
||||||
import org.openjdk.jmh.annotations.Fork;
|
import org.openjdk.jmh.annotations.Fork;
|
||||||
|
@ -52,7 +55,11 @@ public class QueueRWBenchmark {
|
||||||
|
|
||||||
private static final Event EVENT = new Event();
|
private static final Event EVENT = new Event();
|
||||||
|
|
||||||
private Queue queue;
|
private ArrayBlockingQueue<Event> queueArrayBlocking;
|
||||||
|
|
||||||
|
private Queue queuePersisted;
|
||||||
|
|
||||||
|
private Queue queueMemory;
|
||||||
|
|
||||||
private String path;
|
private String path;
|
||||||
|
|
||||||
|
@ -60,21 +67,26 @@ public class QueueRWBenchmark {
|
||||||
|
|
||||||
@Setup
|
@Setup
|
||||||
public void setUp() throws IOException, CloneNotSupportedException {
|
public void setUp() throws IOException, CloneNotSupportedException {
|
||||||
final Settings settings = settings();
|
final Settings settingsPersisted = settings(true);
|
||||||
EVENT.setField("Foo", "Bar");
|
EVENT.setField("Foo", "Bar");
|
||||||
EVENT.setField("Foo1", "Bar1");
|
EVENT.setField("Foo1", "Bar1");
|
||||||
EVENT.setField("Foo2", "Bar2");
|
EVENT.setField("Foo2", "Bar2");
|
||||||
EVENT.setField("Foo3", "Bar3");
|
EVENT.setField("Foo3", "Bar3");
|
||||||
EVENT.setField("Foo4", "Bar4");
|
EVENT.setField("Foo4", "Bar4");
|
||||||
path = settings.getDirPath();
|
path = settingsPersisted.getDirPath();
|
||||||
queue = new Queue(settings);
|
queuePersisted = new Queue(settingsPersisted);
|
||||||
queue.open();
|
queueArrayBlocking = new ArrayBlockingQueue<>(ACK_INTERVAL);
|
||||||
|
queueMemory = new Queue(settings(false));
|
||||||
|
queuePersisted.open();
|
||||||
|
queueMemory.open();
|
||||||
exec = Executors.newSingleThreadExecutor();
|
exec = Executors.newSingleThreadExecutor();
|
||||||
}
|
}
|
||||||
|
|
||||||
@TearDown
|
@TearDown
|
||||||
public void tearDown() throws IOException {
|
public void tearDown() throws IOException {
|
||||||
queue.close();
|
queuePersisted.close();
|
||||||
|
queueMemory.close();
|
||||||
|
queueArrayBlocking.clear();
|
||||||
FileUtils.deleteDirectory(new File(path));
|
FileUtils.deleteDirectory(new File(path));
|
||||||
exec.shutdownNow();
|
exec.shutdownNow();
|
||||||
}
|
}
|
||||||
|
@ -85,16 +97,36 @@ public class QueueRWBenchmark {
|
||||||
final Future<?> future = exec.submit(() -> {
|
final Future<?> future = exec.submit(() -> {
|
||||||
for (int i = 0; i < EVENTS_PER_INVOCATION; ++i) {
|
for (int i = 0; i < EVENTS_PER_INVOCATION; ++i) {
|
||||||
try {
|
try {
|
||||||
final Event evnt = EVENT.clone();
|
this.queuePersisted.write(EVENT);
|
||||||
evnt.setTimestamp(Timestamp.now());
|
} catch (final IOException ex) {
|
||||||
this.queue.write(evnt);
|
|
||||||
} catch (final IOException | CloneNotSupportedException ex) {
|
|
||||||
throw new IllegalStateException(ex);
|
throw new IllegalStateException(ex);
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
});
|
});
|
||||||
for (int i = 0; i < EVENTS_PER_INVOCATION / BATCH_SIZE; ++i) {
|
for (int i = 0; i < EVENTS_PER_INVOCATION / BATCH_SIZE; ++i) {
|
||||||
try (Batch batch = queue.readBatch(BATCH_SIZE)) {
|
try (Batch batch = queuePersisted.readBatch(BATCH_SIZE)) {
|
||||||
|
for (final Queueable elem : batch.getElements()) {
|
||||||
|
blackhole.consume(elem);
|
||||||
|
}
|
||||||
|
}
|
||||||
|
}
|
||||||
|
future.get();
|
||||||
|
}
|
||||||
|
|
||||||
|
@Benchmark
|
||||||
|
@OperationsPerInvocation(EVENTS_PER_INVOCATION)
|
||||||
|
public final void readFromMemoryQueue(final Blackhole blackhole) throws Exception {
|
||||||
|
final Future<?> future = exec.submit(() -> {
|
||||||
|
for (int i = 0; i < EVENTS_PER_INVOCATION; ++i) {
|
||||||
|
try {
|
||||||
|
this.queueMemory.write(EVENT);
|
||||||
|
} catch (final IOException ex) {
|
||||||
|
throw new IllegalStateException(ex);
|
||||||
|
}
|
||||||
|
}
|
||||||
|
});
|
||||||
|
for (int i = 0; i < EVENTS_PER_INVOCATION / BATCH_SIZE; ++i) {
|
||||||
|
try (Batch batch = queueMemory.readBatch(BATCH_SIZE)) {
|
||||||
for (final Queueable elem : batch.getElements()) {
|
for (final Queueable elem : batch.getElements()) {
|
||||||
blackhole.consume(elem);
|
blackhole.consume(elem);
|
||||||
}
|
}
|
||||||
|
@ -106,20 +138,17 @@ public class QueueRWBenchmark {
|
||||||
@Benchmark
|
@Benchmark
|
||||||
@OperationsPerInvocation(EVENTS_PER_INVOCATION)
|
@OperationsPerInvocation(EVENTS_PER_INVOCATION)
|
||||||
public final void readFromArrayBlockingQueue(final Blackhole blackhole) throws Exception {
|
public final void readFromArrayBlockingQueue(final Blackhole blackhole) throws Exception {
|
||||||
final ArrayBlockingQueue<Event> arrayBlockingQueue = new ArrayBlockingQueue<>(ACK_INTERVAL);
|
|
||||||
final Future<?> future = exec.submit(() -> {
|
final Future<?> future = exec.submit(() -> {
|
||||||
for (int i = 0; i < EVENTS_PER_INVOCATION; ++i) {
|
for (int i = 0; i < EVENTS_PER_INVOCATION; ++i) {
|
||||||
try {
|
try {
|
||||||
final Event evnt = EVENT.clone();
|
queueArrayBlocking.put(EVENT);
|
||||||
evnt.setTimestamp(Timestamp.now());
|
} catch (final InterruptedException ex) {
|
||||||
arrayBlockingQueue.put(evnt);
|
|
||||||
} catch (final CloneNotSupportedException | InterruptedException ex) {
|
|
||||||
throw new IllegalStateException(ex);
|
throw new IllegalStateException(ex);
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
});
|
});
|
||||||
for (int i = 0; i < EVENTS_PER_INVOCATION; ++i) {
|
for (int i = 0; i < EVENTS_PER_INVOCATION; ++i) {
|
||||||
blackhole.consume(arrayBlockingQueue.take());
|
blackhole.consume(queueArrayBlocking.take());
|
||||||
}
|
}
|
||||||
future.get();
|
future.get();
|
||||||
}
|
}
|
||||||
|
@ -132,14 +161,23 @@ public class QueueRWBenchmark {
|
||||||
new Runner(opt).run();
|
new Runner(opt).run();
|
||||||
}
|
}
|
||||||
|
|
||||||
private static Settings settings() {
|
private static Settings settings(final boolean persisted) {
|
||||||
|
final PageIOFactory pageIOFactory;
|
||||||
|
final CheckpointIOFactory checkpointIOFactory;
|
||||||
|
if (persisted) {
|
||||||
|
pageIOFactory = MmapPageIO::new;
|
||||||
|
checkpointIOFactory = FileCheckpointIO::new;
|
||||||
|
} else {
|
||||||
|
pageIOFactory = ByteBufferPageIO::new;
|
||||||
|
checkpointIOFactory = MemoryCheckpointIO::new;
|
||||||
|
}
|
||||||
return SettingsImpl.fileSettingsBuilder(Files.createTempDir().getPath())
|
return SettingsImpl.fileSettingsBuilder(Files.createTempDir().getPath())
|
||||||
.capacity(256 * 1024 * 1024)
|
.capacity(256 * 1024 * 1024)
|
||||||
.queueMaxBytes(Long.MAX_VALUE)
|
.queueMaxBytes(Long.MAX_VALUE)
|
||||||
.elementIOFactory(MmapPageIO::new)
|
.elementIOFactory(pageIOFactory)
|
||||||
.checkpointMaxWrites(ACK_INTERVAL)
|
.checkpointMaxWrites(ACK_INTERVAL)
|
||||||
.checkpointMaxAcks(ACK_INTERVAL)
|
.checkpointMaxAcks(ACK_INTERVAL)
|
||||||
.checkpointIOFactory(FileCheckpointIO::new)
|
.checkpointIOFactory(checkpointIOFactory)
|
||||||
.elementClass(Event.class).build();
|
.elementClass(Event.class).build();
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
Loading…
Add table
Add a link
Reference in a new issue