mirror of https://github.com/apache/druid.git
Remove locks from StupidPool
This commit is contained in:
parent
9dd18de1a5
commit
7fe8562980
|
@ -56,7 +56,7 @@
|
|||
<properties>
|
||||
<project.build.sourceEncoding>UTF-8</project.build.sourceEncoding>
|
||||
<jmh.version>1.9.2</jmh.version>
|
||||
<javac.target>1.6</javac.target>
|
||||
<javac.target>1.7</javac.target>
|
||||
<uberjar.name>benchmarks</uberjar.name>
|
||||
</properties>
|
||||
|
||||
|
|
|
@ -0,0 +1,90 @@
|
|||
/*
|
||||
* Licensed to Metamarkets Group Inc. (Metamarkets) under one
|
||||
* or more contributor license agreements. See the NOTICE file
|
||||
* distributed with this work for additional information
|
||||
* regarding copyright ownership. Metamarkets 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 io.druid.benchmark;
|
||||
|
||||
import com.google.common.base.Supplier;
|
||||
import com.google.common.io.Closer;
|
||||
import com.metamx.common.logger.Logger;
|
||||
import io.druid.collections.ResourceHolder;
|
||||
import io.druid.collections.StupidPool;
|
||||
import org.openjdk.jmh.annotations.Benchmark;
|
||||
import org.openjdk.jmh.annotations.BenchmarkMode;
|
||||
import org.openjdk.jmh.annotations.Level;
|
||||
import org.openjdk.jmh.annotations.Mode;
|
||||
import org.openjdk.jmh.annotations.OperationsPerInvocation;
|
||||
import org.openjdk.jmh.annotations.OutputTimeUnit;
|
||||
import org.openjdk.jmh.annotations.Scope;
|
||||
import org.openjdk.jmh.annotations.Setup;
|
||||
import org.openjdk.jmh.annotations.State;
|
||||
import org.openjdk.jmh.annotations.TearDown;
|
||||
import org.openjdk.jmh.annotations.Threads;
|
||||
import org.openjdk.jmh.infra.Blackhole;
|
||||
|
||||
import java.io.IOException;
|
||||
import java.util.concurrent.TimeUnit;
|
||||
import java.util.concurrent.atomic.AtomicLong;
|
||||
|
||||
@State(Scope.Benchmark)
|
||||
public class StupidPoolConcurrencyBenchmark
|
||||
{
|
||||
private static final Logger log = new Logger(StupidPoolConcurrencyBenchmark.class);
|
||||
|
||||
|
||||
@Setup(Level.Iteration)
|
||||
public void setup() throws IOException
|
||||
{
|
||||
|
||||
}
|
||||
|
||||
@TearDown(Level.Iteration)
|
||||
public void teardown()
|
||||
{
|
||||
|
||||
}
|
||||
|
||||
private static final Object simpleObject = new Object();
|
||||
|
||||
@State(Scope.Benchmark)
|
||||
public static class BenchmarkPool
|
||||
{
|
||||
private final AtomicLong numPools = new AtomicLong(0L);
|
||||
private final StupidPool<Object> pool = new StupidPool<>(
|
||||
new Supplier<Object>()
|
||||
{
|
||||
@Override
|
||||
public Object get()
|
||||
{
|
||||
numPools.incrementAndGet();
|
||||
return simpleObject;
|
||||
}
|
||||
}
|
||||
);
|
||||
}
|
||||
|
||||
@Benchmark
|
||||
@BenchmarkMode(Mode.Throughput)
|
||||
@OutputTimeUnit(TimeUnit.MICROSECONDS)
|
||||
public void hammerQueue(BenchmarkPool pool, Blackhole blackhole) throws IOException
|
||||
{
|
||||
try(ResourceHolder<Object> holder = pool.pool.take()){
|
||||
blackhole.consume(holder);
|
||||
}
|
||||
}
|
||||
}
|
|
@ -18,12 +18,13 @@
|
|||
package io.druid.collections;
|
||||
|
||||
import com.google.common.base.Supplier;
|
||||
import com.google.common.collect.Lists;
|
||||
import com.metamx.common.ISE;
|
||||
import com.metamx.common.logger.Logger;
|
||||
|
||||
import java.io.IOException;
|
||||
import java.util.LinkedList;
|
||||
import java.util.Queue;
|
||||
import java.util.concurrent.ConcurrentLinkedQueue;
|
||||
import java.util.concurrent.atomic.AtomicBoolean;
|
||||
|
||||
/**
|
||||
*/
|
||||
|
@ -33,7 +34,7 @@ public class StupidPool<T>
|
|||
|
||||
private final Supplier<T> generator;
|
||||
|
||||
private final LinkedList<T> objects = Lists.newLinkedList();
|
||||
private final Queue<T> objects = new ConcurrentLinkedQueue<>();
|
||||
|
||||
public StupidPool(
|
||||
Supplier<T> generator
|
||||
|
@ -44,18 +45,13 @@ public class StupidPool<T>
|
|||
|
||||
public ResourceHolder<T> take()
|
||||
{
|
||||
synchronized (objects) {
|
||||
if (objects.size() > 0) {
|
||||
return new ObjectResourceHolder(objects.removeFirst());
|
||||
}
|
||||
}
|
||||
|
||||
return new ObjectResourceHolder(generator.get());
|
||||
final T obj = objects.poll();
|
||||
return obj == null ? new ObjectResourceHolder(generator.get()) : new ObjectResourceHolder(obj);
|
||||
}
|
||||
|
||||
private class ObjectResourceHolder implements ResourceHolder<T>
|
||||
{
|
||||
private boolean closed = false;
|
||||
private AtomicBoolean closed = new AtomicBoolean(false);
|
||||
private final T object;
|
||||
|
||||
public ObjectResourceHolder(final T object)
|
||||
|
@ -63,10 +59,12 @@ public class StupidPool<T>
|
|||
this.object = object;
|
||||
}
|
||||
|
||||
// WARNING: it is entirely possible for a caller to hold onto the object and call ObjectResourceHolder.close,
|
||||
// Then still use that object even though it will be offered to someone else in StupidPool.take
|
||||
@Override
|
||||
public synchronized T get()
|
||||
public T get()
|
||||
{
|
||||
if (closed) {
|
||||
if (closed.get()) {
|
||||
throw new ISE("Already Closed!");
|
||||
}
|
||||
|
||||
|
@ -74,24 +72,27 @@ public class StupidPool<T>
|
|||
}
|
||||
|
||||
@Override
|
||||
public synchronized void close() throws IOException
|
||||
public void close() throws IOException
|
||||
{
|
||||
if (closed) {
|
||||
if (!closed.compareAndSet(false, true)) {
|
||||
log.warn(new ISE("Already Closed!"), "Already closed");
|
||||
return;
|
||||
}
|
||||
|
||||
synchronized (objects) {
|
||||
closed = true;
|
||||
objects.addLast(object);
|
||||
if (!objects.offer(object)) {
|
||||
log.warn(new ISE("Queue offer failed"), "Could not offer object [%s] back into the queue", object);
|
||||
}
|
||||
}
|
||||
|
||||
@Override
|
||||
protected void finalize() throws Throwable
|
||||
{
|
||||
if (! closed) {
|
||||
log.warn("Not closed! Object was[%s]. Allowing gc to prevent leak.", object);
|
||||
try {
|
||||
if (!closed.get()) {
|
||||
log.warn("Not closed! Object was[%s]. Allowing gc to prevent leak.", object);
|
||||
}
|
||||
}
|
||||
finally {
|
||||
super.finalize();
|
||||
}
|
||||
}
|
||||
}
|
||||
|
|
Loading…
Reference in New Issue