mirror of https://github.com/apache/druid.git
Merge pull request #1744 from metamx/memcached-connection-pooling
Memcached connection pooling
This commit is contained in:
commit
d325bb55ae
|
@ -0,0 +1,146 @@
|
|||
/*
|
||||
* 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.collections;
|
||||
|
||||
import com.google.common.base.Preconditions;
|
||||
import com.google.common.base.Supplier;
|
||||
import com.google.common.base.Throwables;
|
||||
import com.metamx.common.logger.Logger;
|
||||
|
||||
import java.io.IOException;
|
||||
import java.util.concurrent.PriorityBlockingQueue;
|
||||
import java.util.concurrent.atomic.AtomicInteger;
|
||||
|
||||
/**
|
||||
* Simple load balancing pool that always returns the least used item.
|
||||
*
|
||||
* An item's usage is incremented every time one gets requested from the pool
|
||||
* and is decremented every time close is called on the holder.
|
||||
*
|
||||
* The pool eagerly instantiates all the items in the pool when created,
|
||||
* using the given supplier.
|
||||
*
|
||||
* @param <T> type of items to pool
|
||||
*/
|
||||
public class LoadBalancingPool<T> implements Supplier<ResourceHolder<T>>
|
||||
{
|
||||
private static final Logger log = new Logger(LoadBalancingPool.class);
|
||||
|
||||
private final Supplier<T> generator;
|
||||
private final int capacity;
|
||||
private final PriorityBlockingQueue<CountingHolder> queue;
|
||||
|
||||
public LoadBalancingPool(int capacity, Supplier<T> generator)
|
||||
{
|
||||
Preconditions.checkArgument(capacity > 0, "capacity must be greater than 0");
|
||||
Preconditions.checkNotNull(generator);
|
||||
|
||||
this.generator = generator;
|
||||
this.capacity = capacity;
|
||||
this.queue = new PriorityBlockingQueue<>(capacity);
|
||||
|
||||
// eagerly intantiate all items in the pool
|
||||
init();
|
||||
}
|
||||
|
||||
private void init() {
|
||||
for(int i = 0; i < capacity; ++i) {
|
||||
queue.offer(new CountingHolder(generator.get()));
|
||||
}
|
||||
}
|
||||
|
||||
public ResourceHolder<T> get()
|
||||
{
|
||||
final CountingHolder holder;
|
||||
// items never stay out of the queue for long, so we'll get one eventually
|
||||
try {
|
||||
holder = queue.take();
|
||||
} catch(InterruptedException e) {
|
||||
Thread.currentThread().interrupt();
|
||||
throw Throwables.propagate(e);
|
||||
}
|
||||
|
||||
// synchronize on item to ensure count cannot get changed by
|
||||
// CountingHolder.close right after we put the item back in the queue
|
||||
synchronized (holder) {
|
||||
holder.count.incrementAndGet();
|
||||
queue.offer(holder);
|
||||
}
|
||||
return holder;
|
||||
}
|
||||
|
||||
private class CountingHolder implements ResourceHolder<T>, Comparable<CountingHolder>
|
||||
{
|
||||
private AtomicInteger count = new AtomicInteger(0);
|
||||
private final T object;
|
||||
|
||||
public CountingHolder(final T object)
|
||||
{
|
||||
this.object = object;
|
||||
}
|
||||
|
||||
@Override
|
||||
public T get()
|
||||
{
|
||||
return object;
|
||||
}
|
||||
|
||||
/**
|
||||
* Not idempotent, should only be called once when done using the resource
|
||||
*
|
||||
* @throws IOException
|
||||
*/
|
||||
@Override
|
||||
public void close() throws IOException
|
||||
{
|
||||
// ensures count always gets adjusted while item is removed from the queue
|
||||
synchronized (this) {
|
||||
// item may not be in queue if another thread is calling LoadBalancingPool.get()
|
||||
// at the same time; in that case let the other thread put it back.
|
||||
boolean removed = queue.remove(this);
|
||||
count.decrementAndGet();
|
||||
if (removed) {
|
||||
queue.offer(this);
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
@Override
|
||||
public int compareTo(CountingHolder o)
|
||||
{
|
||||
return Integer.compare(count.get(), o.count.get());
|
||||
}
|
||||
|
||||
|
||||
@Override
|
||||
protected void finalize() throws Throwable
|
||||
{
|
||||
try {
|
||||
final int shouldBeZero = count.get();
|
||||
if (shouldBeZero != 0) {
|
||||
log.warn("Expected 0 resource count, got [%d]! Object was[%s].", shouldBeZero, object);
|
||||
}
|
||||
}
|
||||
finally {
|
||||
super.finalize();
|
||||
}
|
||||
}
|
||||
}
|
||||
}
|
|
@ -23,5 +23,5 @@ import java.io.Closeable;
|
|||
*/
|
||||
public interface ResourceHolder<T> extends Closeable
|
||||
{
|
||||
public T get();
|
||||
T get();
|
||||
}
|
||||
|
|
|
@ -109,3 +109,4 @@ You can optionally only configure caching to be enabled on the broker by setting
|
|||
|`druid.cache.hosts`|Command separated list of Memcached hosts `<host:port>`.|none|
|
||||
|`druid.cache.maxObjectSize`|Maximum object size in bytes for a Memcached object.|52428800 (50 MB)|
|
||||
|`druid.cache.memcachedPrefix`|Key prefix for all keys in Memcached.|druid|
|
||||
|`druid.cache.numConnections`|Number of memcached connections to use.|1|
|
||||
|
|
|
@ -21,6 +21,8 @@ import com.google.common.base.Charsets;
|
|||
import com.google.common.base.Function;
|
||||
import com.google.common.base.Preconditions;
|
||||
import com.google.common.base.Predicate;
|
||||
import com.google.common.base.Supplier;
|
||||
import com.google.common.base.Suppliers;
|
||||
import com.google.common.base.Throwables;
|
||||
import com.google.common.collect.ImmutableMap;
|
||||
import com.google.common.collect.Maps;
|
||||
|
@ -32,7 +34,11 @@ import com.metamx.emitter.service.ServiceEmitter;
|
|||
import com.metamx.emitter.service.ServiceMetricEvent;
|
||||
import com.metamx.metrics.AbstractMonitor;
|
||||
import com.metamx.metrics.MonitorScheduler;
|
||||
import io.druid.collections.LoadBalancingPool;
|
||||
import io.druid.collections.ResourceHolder;
|
||||
import io.druid.collections.StupidResourceHolder;
|
||||
import net.spy.memcached.AddrUtil;
|
||||
import net.spy.memcached.ConnectionFactory;
|
||||
import net.spy.memcached.ConnectionFactoryBuilder;
|
||||
import net.spy.memcached.FailureMode;
|
||||
import net.spy.memcached.HashAlgorithm;
|
||||
|
@ -40,16 +46,17 @@ import net.spy.memcached.MemcachedClient;
|
|||
import net.spy.memcached.MemcachedClientIF;
|
||||
import net.spy.memcached.internal.BulkFuture;
|
||||
import net.spy.memcached.metrics.MetricCollector;
|
||||
import net.spy.memcached.metrics.MetricType;
|
||||
import net.spy.memcached.ops.LinkedOperationQueueFactory;
|
||||
import net.spy.memcached.ops.OperationQueueFactory;
|
||||
import org.apache.commons.codec.digest.DigestUtils;
|
||||
|
||||
import javax.annotation.Nullable;
|
||||
import java.io.IOException;
|
||||
import java.net.InetSocketAddress;
|
||||
import java.nio.ByteBuffer;
|
||||
import java.util.Arrays;
|
||||
import java.util.HashMap;
|
||||
import java.util.List;
|
||||
import java.util.Map;
|
||||
import java.util.concurrent.ConcurrentHashMap;
|
||||
import java.util.concurrent.ConcurrentMap;
|
||||
|
@ -154,184 +161,206 @@ public class MemcachedCache implements Cache
|
|||
}
|
||||
};
|
||||
|
||||
return new MemcachedCache(
|
||||
new MemcachedClient(
|
||||
new MemcachedCustomConnectionFactoryBuilder()
|
||||
// 1000 repetitions gives us good distribution with murmur3_128
|
||||
// (approx < 5% difference in counts across nodes, with 5 cache nodes)
|
||||
.setKetamaNodeRepetitions(1000)
|
||||
.setHashAlg(MURMUR3_128)
|
||||
.setProtocol(ConnectionFactoryBuilder.Protocol.BINARY)
|
||||
.setLocatorType(ConnectionFactoryBuilder.Locator.CONSISTENT)
|
||||
.setDaemon(true)
|
||||
.setFailureMode(FailureMode.Cancel)
|
||||
.setTranscoder(transcoder)
|
||||
.setShouldOptimize(true)
|
||||
.setOpQueueMaxBlockTime(config.getTimeout())
|
||||
.setOpTimeout(config.getTimeout())
|
||||
.setReadBufferSize(config.getReadBufferSize())
|
||||
.setOpQueueFactory(opQueueFactory)
|
||||
.setEnableMetrics(MetricType.DEBUG) // Not as scary as it sounds
|
||||
.setWriteOpQueueFactory(opQueueFactory)
|
||||
.setReadOpQueueFactory(opQueueFactory)
|
||||
.setMetricCollector(
|
||||
new MetricCollector()
|
||||
{
|
||||
@Override
|
||||
public void addCounter(String name)
|
||||
{
|
||||
if (!interesting.apply(name)) {
|
||||
return;
|
||||
}
|
||||
counters.put(name, new AtomicLong(0L));
|
||||
final MetricCollector metricCollector = new MetricCollector()
|
||||
{
|
||||
@Override
|
||||
public void addCounter(String name)
|
||||
{
|
||||
if (!interesting.apply(name)) {
|
||||
return;
|
||||
}
|
||||
counters.put(name, new AtomicLong(0L));
|
||||
|
||||
if (log.isDebugEnabled()) {
|
||||
log.debug("Add Counter [%s]", name);
|
||||
}
|
||||
}
|
||||
if (log.isDebugEnabled()) {
|
||||
log.debug("Add Counter [%s]", name);
|
||||
}
|
||||
}
|
||||
|
||||
@Override
|
||||
public void removeCounter(String name)
|
||||
{
|
||||
if (!interesting.apply(name)) {
|
||||
return;
|
||||
}
|
||||
counters.remove(name);
|
||||
@Override
|
||||
public void removeCounter(String name)
|
||||
{
|
||||
if (!interesting.apply(name)) {
|
||||
return;
|
||||
}
|
||||
counters.remove(name);
|
||||
|
||||
if (log.isDebugEnabled()) {
|
||||
log.debug("Remove Counter [%s]", name);
|
||||
}
|
||||
}
|
||||
if (log.isDebugEnabled()) {
|
||||
log.debug("Remove Counter [%s]", name);
|
||||
}
|
||||
}
|
||||
|
||||
@Override
|
||||
public void incrementCounter(String name)
|
||||
{
|
||||
if (!interesting.apply(name)) {
|
||||
return;
|
||||
}
|
||||
AtomicLong counter = counters.get(name);
|
||||
if (counter == null) {
|
||||
counters.putIfAbsent(name, new AtomicLong(0));
|
||||
counter = counters.get(name);
|
||||
}
|
||||
counter.incrementAndGet();
|
||||
@Override
|
||||
public void incrementCounter(String name)
|
||||
{
|
||||
if (!interesting.apply(name)) {
|
||||
return;
|
||||
}
|
||||
AtomicLong counter = counters.get(name);
|
||||
if (counter == null) {
|
||||
counters.putIfAbsent(name, new AtomicLong(0));
|
||||
counter = counters.get(name);
|
||||
}
|
||||
counter.incrementAndGet();
|
||||
|
||||
if (log.isDebugEnabled()) {
|
||||
log.debug("Incrament [%s]", name);
|
||||
}
|
||||
}
|
||||
if (log.isDebugEnabled()) {
|
||||
log.debug("Increment [%s]", name);
|
||||
}
|
||||
}
|
||||
|
||||
@Override
|
||||
public void incrementCounter(String name, int amount)
|
||||
{
|
||||
if (!interesting.apply(name)) {
|
||||
return;
|
||||
}
|
||||
AtomicLong counter = counters.get(name);
|
||||
if (counter == null) {
|
||||
counters.putIfAbsent(name, new AtomicLong(0));
|
||||
counter = counters.get(name);
|
||||
}
|
||||
counter.addAndGet(amount);
|
||||
@Override
|
||||
public void incrementCounter(String name, int amount)
|
||||
{
|
||||
if (!interesting.apply(name)) {
|
||||
return;
|
||||
}
|
||||
AtomicLong counter = counters.get(name);
|
||||
if (counter == null) {
|
||||
counters.putIfAbsent(name, new AtomicLong(0));
|
||||
counter = counters.get(name);
|
||||
}
|
||||
counter.addAndGet(amount);
|
||||
|
||||
if (log.isDebugEnabled()) {
|
||||
log.debug("Increment [%s] %d", name, amount);
|
||||
}
|
||||
}
|
||||
if (log.isDebugEnabled()) {
|
||||
log.debug("Increment [%s] %d", name, amount);
|
||||
}
|
||||
}
|
||||
|
||||
@Override
|
||||
public void decrementCounter(String name)
|
||||
{
|
||||
if (!interesting.apply(name)) {
|
||||
return;
|
||||
}
|
||||
AtomicLong counter = counters.get(name);
|
||||
if (counter == null) {
|
||||
counters.putIfAbsent(name, new AtomicLong(0));
|
||||
counter = counters.get(name);
|
||||
}
|
||||
counter.decrementAndGet();
|
||||
@Override
|
||||
public void decrementCounter(String name)
|
||||
{
|
||||
if (!interesting.apply(name)) {
|
||||
return;
|
||||
}
|
||||
AtomicLong counter = counters.get(name);
|
||||
if (counter == null) {
|
||||
counters.putIfAbsent(name, new AtomicLong(0));
|
||||
counter = counters.get(name);
|
||||
}
|
||||
counter.decrementAndGet();
|
||||
|
||||
if (log.isDebugEnabled()) {
|
||||
log.debug("Decrement [%s]", name);
|
||||
}
|
||||
}
|
||||
if (log.isDebugEnabled()) {
|
||||
log.debug("Decrement [%s]", name);
|
||||
}
|
||||
}
|
||||
|
||||
@Override
|
||||
public void decrementCounter(String name, int amount)
|
||||
{
|
||||
if (!interesting.apply(name)) {
|
||||
return;
|
||||
}
|
||||
AtomicLong counter = counters.get(name);
|
||||
if (counter == null) {
|
||||
counters.putIfAbsent(name, new AtomicLong(0L));
|
||||
counter = counters.get(name);
|
||||
}
|
||||
counter.addAndGet(-amount);
|
||||
@Override
|
||||
public void decrementCounter(String name, int amount)
|
||||
{
|
||||
if (!interesting.apply(name)) {
|
||||
return;
|
||||
}
|
||||
AtomicLong counter = counters.get(name);
|
||||
if (counter == null) {
|
||||
counters.putIfAbsent(name, new AtomicLong(0L));
|
||||
counter = counters.get(name);
|
||||
}
|
||||
counter.addAndGet(-amount);
|
||||
|
||||
if (log.isDebugEnabled()) {
|
||||
log.debug("Decrement [%s] %d", name, amount);
|
||||
}
|
||||
}
|
||||
if (log.isDebugEnabled()) {
|
||||
log.debug("Decrement [%s] %d", name, amount);
|
||||
}
|
||||
}
|
||||
|
||||
@Override
|
||||
public void addMeter(String name)
|
||||
{
|
||||
meters.put(name, new AtomicLong(0L));
|
||||
if (log.isDebugEnabled()) {
|
||||
log.debug("Adding meter [%s]", name);
|
||||
}
|
||||
}
|
||||
@Override
|
||||
public void addMeter(String name)
|
||||
{
|
||||
meters.put(name, new AtomicLong(0L));
|
||||
if (log.isDebugEnabled()) {
|
||||
log.debug("Adding meter [%s]", name);
|
||||
}
|
||||
}
|
||||
|
||||
@Override
|
||||
public void removeMeter(String name)
|
||||
{
|
||||
meters.remove(name);
|
||||
if (log.isDebugEnabled()) {
|
||||
log.debug("Removing meter [%s]", name);
|
||||
}
|
||||
}
|
||||
@Override
|
||||
public void removeMeter(String name)
|
||||
{
|
||||
meters.remove(name);
|
||||
if (log.isDebugEnabled()) {
|
||||
log.debug("Removing meter [%s]", name);
|
||||
}
|
||||
}
|
||||
|
||||
@Override
|
||||
public void markMeter(String name)
|
||||
{
|
||||
AtomicLong meter = meters.get(name);
|
||||
if (meter == null) {
|
||||
meters.putIfAbsent(name, new AtomicLong(0L));
|
||||
meter = meters.get(name);
|
||||
}
|
||||
meter.incrementAndGet();
|
||||
@Override
|
||||
public void markMeter(String name)
|
||||
{
|
||||
AtomicLong meter = meters.get(name);
|
||||
if (meter == null) {
|
||||
meters.putIfAbsent(name, new AtomicLong(0L));
|
||||
meter = meters.get(name);
|
||||
}
|
||||
meter.incrementAndGet();
|
||||
|
||||
if (log.isDebugEnabled()) {
|
||||
log.debug("Increment counter [%s]", name);
|
||||
}
|
||||
}
|
||||
if (log.isDebugEnabled()) {
|
||||
log.debug("Increment counter [%s]", name);
|
||||
}
|
||||
}
|
||||
|
||||
@Override
|
||||
public void addHistogram(String name)
|
||||
{
|
||||
log.debug("Ignoring add histogram [%s]", name);
|
||||
}
|
||||
@Override
|
||||
public void addHistogram(String name)
|
||||
{
|
||||
log.debug("Ignoring add histogram [%s]", name);
|
||||
}
|
||||
|
||||
@Override
|
||||
public void removeHistogram(String name)
|
||||
{
|
||||
log.debug("Ignoring remove histogram [%s]", name);
|
||||
}
|
||||
@Override
|
||||
public void removeHistogram(String name)
|
||||
{
|
||||
log.debug("Ignoring remove histogram [%s]", name);
|
||||
}
|
||||
|
||||
@Override
|
||||
public void updateHistogram(String name, int amount)
|
||||
{
|
||||
log.debug("Ignoring update histogram [%s]: %d", name, amount);
|
||||
}
|
||||
}
|
||||
)
|
||||
.build(),
|
||||
AddrUtil.getAddresses(config.getHosts())
|
||||
),
|
||||
config
|
||||
);
|
||||
@Override
|
||||
public void updateHistogram(String name, int amount)
|
||||
{
|
||||
log.debug("Ignoring update histogram [%s]: %d", name, amount);
|
||||
}
|
||||
};
|
||||
|
||||
final ConnectionFactory connectionFactory = new MemcachedCustomConnectionFactoryBuilder()
|
||||
// 1000 repetitions gives us good distribution with murmur3_128
|
||||
// (approx < 5% difference in counts across nodes, with 5 cache nodes)
|
||||
.setKetamaNodeRepetitions(1000)
|
||||
.setHashAlg(MURMUR3_128)
|
||||
.setProtocol(ConnectionFactoryBuilder.Protocol.BINARY)
|
||||
.setLocatorType(ConnectionFactoryBuilder.Locator.CONSISTENT)
|
||||
.setDaemon(true)
|
||||
.setFailureMode(FailureMode.Cancel)
|
||||
.setTranscoder(transcoder)
|
||||
.setShouldOptimize(true)
|
||||
.setOpQueueMaxBlockTime(config.getTimeout())
|
||||
.setOpTimeout(config.getTimeout())
|
||||
.setReadBufferSize(config.getReadBufferSize())
|
||||
.setOpQueueFactory(opQueueFactory)
|
||||
.setMetricCollector(metricCollector)
|
||||
.build();
|
||||
|
||||
final List<InetSocketAddress> hosts = AddrUtil.getAddresses(config.getHosts());
|
||||
|
||||
|
||||
final Supplier<ResourceHolder<MemcachedClientIF>> clientSupplier;
|
||||
|
||||
if (config.getNumConnections() > 1) {
|
||||
clientSupplier = new LoadBalancingPool<MemcachedClientIF>(
|
||||
config.getNumConnections(),
|
||||
new Supplier<MemcachedClientIF>()
|
||||
{
|
||||
@Override
|
||||
public MemcachedClientIF get()
|
||||
{
|
||||
try {
|
||||
return new MemcachedClient(connectionFactory, hosts);
|
||||
}
|
||||
catch (IOException e) {
|
||||
log.error(e, "Unable to create memcached client");
|
||||
throw Throwables.propagate(e);
|
||||
}
|
||||
}
|
||||
}
|
||||
);
|
||||
} else {
|
||||
clientSupplier = Suppliers.<ResourceHolder<MemcachedClientIF>>ofInstance(
|
||||
StupidResourceHolder.<MemcachedClientIF>create(new MemcachedClient(connectionFactory, hosts))
|
||||
);
|
||||
}
|
||||
|
||||
return new MemcachedCache(clientSupplier, config);
|
||||
}
|
||||
catch (IOException e) {
|
||||
throw Throwables.propagate(e);
|
||||
|
@ -342,7 +371,7 @@ public class MemcachedCache implements Cache
|
|||
private final int expiration;
|
||||
private final String memcachedPrefix;
|
||||
|
||||
private final MemcachedClientIF client;
|
||||
private final Supplier<ResourceHolder<MemcachedClientIF>> client;
|
||||
|
||||
private final AtomicLong hitCount = new AtomicLong(0);
|
||||
private final AtomicLong missCount = new AtomicLong(0);
|
||||
|
@ -350,7 +379,7 @@ public class MemcachedCache implements Cache
|
|||
private final AtomicLong errorCount = new AtomicLong(0);
|
||||
|
||||
|
||||
MemcachedCache(MemcachedClientIF client, MemcachedCacheConfig config)
|
||||
MemcachedCache(Supplier<ResourceHolder<MemcachedClientIF>> client, MemcachedCacheConfig config)
|
||||
{
|
||||
Preconditions.checkArgument(
|
||||
config.getMemcachedPrefix().length() <= MAX_PREFIX_LENGTH,
|
||||
|
@ -381,52 +410,64 @@ public class MemcachedCache implements Cache
|
|||
@Override
|
||||
public byte[] get(NamedKey key)
|
||||
{
|
||||
Future<Object> future;
|
||||
try {
|
||||
future = client.asyncGet(computeKeyHash(memcachedPrefix, key));
|
||||
}
|
||||
catch (IllegalStateException e) {
|
||||
// operation did not get queued in time (queue is full)
|
||||
errorCount.incrementAndGet();
|
||||
log.warn(e, "Unable to queue cache operation");
|
||||
return null;
|
||||
}
|
||||
try {
|
||||
byte[] bytes = (byte[]) future.get(timeout, TimeUnit.MILLISECONDS);
|
||||
if (bytes != null) {
|
||||
hitCount.incrementAndGet();
|
||||
} else {
|
||||
missCount.incrementAndGet();
|
||||
try (ResourceHolder<MemcachedClientIF> clientHolder = client.get()) {
|
||||
Future<Object> future;
|
||||
try {
|
||||
future = clientHolder.get().asyncGet(computeKeyHash(memcachedPrefix, key));
|
||||
}
|
||||
catch (IllegalStateException e) {
|
||||
// operation did not get queued in time (queue is full)
|
||||
errorCount.incrementAndGet();
|
||||
log.warn(e, "Unable to queue cache operation");
|
||||
return null;
|
||||
}
|
||||
try {
|
||||
byte[] bytes = (byte[]) future.get(timeout, TimeUnit.MILLISECONDS);
|
||||
if (bytes != null) {
|
||||
hitCount.incrementAndGet();
|
||||
} else {
|
||||
missCount.incrementAndGet();
|
||||
}
|
||||
return bytes == null ? null : deserializeValue(key, bytes);
|
||||
}
|
||||
catch (TimeoutException e) {
|
||||
timeoutCount.incrementAndGet();
|
||||
future.cancel(false);
|
||||
return null;
|
||||
}
|
||||
catch (InterruptedException e) {
|
||||
Thread.currentThread().interrupt();
|
||||
throw Throwables.propagate(e);
|
||||
}
|
||||
catch (ExecutionException e) {
|
||||
errorCount.incrementAndGet();
|
||||
log.warn(e, "Exception pulling item from cache");
|
||||
return null;
|
||||
}
|
||||
return bytes == null ? null : deserializeValue(key, bytes);
|
||||
}
|
||||
catch (TimeoutException e) {
|
||||
timeoutCount.incrementAndGet();
|
||||
future.cancel(false);
|
||||
return null;
|
||||
}
|
||||
catch (InterruptedException e) {
|
||||
Thread.currentThread().interrupt();
|
||||
catch (IOException e) {
|
||||
throw Throwables.propagate(e);
|
||||
}
|
||||
catch (ExecutionException e) {
|
||||
errorCount.incrementAndGet();
|
||||
log.warn(e, "Exception pulling item from cache");
|
||||
return null;
|
||||
}
|
||||
}
|
||||
|
||||
@Override
|
||||
public void put(NamedKey key, byte[] value)
|
||||
{
|
||||
try {
|
||||
client.set(computeKeyHash(memcachedPrefix, key), expiration, serializeValue(key, value));
|
||||
try (final ResourceHolder<MemcachedClientIF> clientHolder = client.get()) {
|
||||
clientHolder.get().set(
|
||||
computeKeyHash(memcachedPrefix, key),
|
||||
expiration,
|
||||
serializeValue(key, value)
|
||||
);
|
||||
}
|
||||
catch (IllegalStateException e) {
|
||||
// operation did not get queued in time (queue is full)
|
||||
errorCount.incrementAndGet();
|
||||
log.warn(e, "Unable to queue cache operation");
|
||||
}
|
||||
catch (IOException e) {
|
||||
Throwables.propagate(e);
|
||||
}
|
||||
}
|
||||
|
||||
private static byte[] serializeValue(NamedKey key, byte[] value)
|
||||
|
@ -459,63 +500,68 @@ public class MemcachedCache implements Cache
|
|||
@Override
|
||||
public Map<NamedKey, byte[]> getBulk(Iterable<NamedKey> keys)
|
||||
{
|
||||
Map<String, NamedKey> keyLookup = Maps.uniqueIndex(
|
||||
keys,
|
||||
new Function<NamedKey, String>()
|
||||
{
|
||||
@Override
|
||||
public String apply(
|
||||
@Nullable NamedKey input
|
||||
)
|
||||
try (ResourceHolder<MemcachedClientIF> clientHolder = client.get()) {
|
||||
Map<String, NamedKey> keyLookup = Maps.uniqueIndex(
|
||||
keys,
|
||||
new Function<NamedKey, String>()
|
||||
{
|
||||
return computeKeyHash(memcachedPrefix, input);
|
||||
@Override
|
||||
public String apply(
|
||||
@Nullable NamedKey input
|
||||
)
|
||||
{
|
||||
return computeKeyHash(memcachedPrefix, input);
|
||||
}
|
||||
}
|
||||
);
|
||||
|
||||
Map<NamedKey, byte[]> results = Maps.newHashMap();
|
||||
|
||||
BulkFuture<Map<String, Object>> future;
|
||||
try {
|
||||
future = clientHolder.get().asyncGetBulk(keyLookup.keySet());
|
||||
}
|
||||
catch (IllegalStateException e) {
|
||||
// operation did not get queued in time (queue is full)
|
||||
errorCount.incrementAndGet();
|
||||
log.warn(e, "Unable to queue cache operation");
|
||||
return results;
|
||||
}
|
||||
|
||||
try {
|
||||
Map<String, Object> some = future.getSome(timeout, TimeUnit.MILLISECONDS);
|
||||
|
||||
if (future.isTimeout()) {
|
||||
future.cancel(false);
|
||||
timeoutCount.incrementAndGet();
|
||||
}
|
||||
);
|
||||
missCount.addAndGet(keyLookup.size() - some.size());
|
||||
hitCount.addAndGet(some.size());
|
||||
|
||||
Map<NamedKey, byte[]> results = Maps.newHashMap();
|
||||
for (Map.Entry<String, Object> entry : some.entrySet()) {
|
||||
final NamedKey key = keyLookup.get(entry.getKey());
|
||||
final byte[] value = (byte[]) entry.getValue();
|
||||
results.put(
|
||||
key,
|
||||
value == null ? null : deserializeValue(key, value)
|
||||
);
|
||||
}
|
||||
|
||||
BulkFuture<Map<String, Object>> future;
|
||||
try {
|
||||
future = client.asyncGetBulk(keyLookup.keySet());
|
||||
}
|
||||
catch (IllegalStateException e) {
|
||||
// operation did not get queued in time (queue is full)
|
||||
errorCount.incrementAndGet();
|
||||
log.warn(e, "Unable to queue cache operation");
|
||||
return results;
|
||||
}
|
||||
|
||||
try {
|
||||
Map<String, Object> some = future.getSome(timeout, TimeUnit.MILLISECONDS);
|
||||
|
||||
if (future.isTimeout()) {
|
||||
future.cancel(false);
|
||||
timeoutCount.incrementAndGet();
|
||||
return results;
|
||||
}
|
||||
missCount.addAndGet(keyLookup.size() - some.size());
|
||||
hitCount.addAndGet(some.size());
|
||||
|
||||
for (Map.Entry<String, Object> entry : some.entrySet()) {
|
||||
final NamedKey key = keyLookup.get(entry.getKey());
|
||||
final byte[] value = (byte[]) entry.getValue();
|
||||
results.put(
|
||||
key,
|
||||
value == null ? null : deserializeValue(key, value)
|
||||
);
|
||||
catch (InterruptedException e) {
|
||||
Thread.currentThread().interrupt();
|
||||
throw Throwables.propagate(e);
|
||||
}
|
||||
catch (ExecutionException e) {
|
||||
errorCount.incrementAndGet();
|
||||
log.warn(e, "Exception pulling item from cache");
|
||||
return results;
|
||||
}
|
||||
|
||||
return results;
|
||||
}
|
||||
catch (InterruptedException e) {
|
||||
Thread.currentThread().interrupt();
|
||||
catch (IOException e) {
|
||||
throw Throwables.propagate(e);
|
||||
}
|
||||
catch (ExecutionException e) {
|
||||
errorCount.incrementAndGet();
|
||||
log.warn(e, "Exception pulling item from cache");
|
||||
return results;
|
||||
}
|
||||
}
|
||||
|
||||
@Override
|
||||
|
|
|
@ -51,6 +51,10 @@ public class MemcachedCacheConfig
|
|||
@JsonProperty
|
||||
private long maxOperationQueueSize = 0;
|
||||
|
||||
// size of memcached connection pool
|
||||
@JsonProperty
|
||||
private int numConnections = 1;
|
||||
|
||||
public int getExpiration()
|
||||
{
|
||||
return expiration;
|
||||
|
@ -85,4 +89,9 @@ public class MemcachedCacheConfig
|
|||
{
|
||||
return readBufferSize;
|
||||
}
|
||||
|
||||
public int getNumConnections()
|
||||
{
|
||||
return numConnections;
|
||||
}
|
||||
}
|
||||
|
|
|
@ -20,7 +20,10 @@ package io.druid.client.cache;
|
|||
import com.google.caliper.Param;
|
||||
import com.google.caliper.Runner;
|
||||
import com.google.caliper.SimpleBenchmark;
|
||||
import com.google.common.base.Suppliers;
|
||||
import com.google.common.collect.Lists;
|
||||
import io.druid.collections.ResourceHolder;
|
||||
import io.druid.collections.StupidResourceHolder;
|
||||
import net.spy.memcached.AddrUtil;
|
||||
import net.spy.memcached.ConnectionFactoryBuilder;
|
||||
import net.spy.memcached.DefaultHashAlgorithm;
|
||||
|
@ -77,7 +80,9 @@ public class MemcachedCacheBenchmark extends SimpleBenchmark
|
|||
|
||||
|
||||
cache = new MemcachedCache(
|
||||
client,
|
||||
Suppliers.<ResourceHolder<MemcachedClientIF>>ofInstance(
|
||||
StupidResourceHolder.create(client)
|
||||
),
|
||||
new MemcachedCacheConfig()
|
||||
{
|
||||
@Override
|
||||
|
|
|
@ -18,6 +18,7 @@
|
|||
package io.druid.client.cache;
|
||||
|
||||
import com.fasterxml.jackson.databind.ObjectMapper;
|
||||
import com.google.common.base.Suppliers;
|
||||
import com.google.common.collect.ImmutableList;
|
||||
import com.google.common.collect.Lists;
|
||||
import com.google.common.collect.Maps;
|
||||
|
@ -34,6 +35,8 @@ import com.metamx.emitter.core.Event;
|
|||
import com.metamx.emitter.service.ServiceEmitter;
|
||||
import com.metamx.metrics.Monitor;
|
||||
import com.metamx.metrics.MonitorScheduler;
|
||||
import io.druid.collections.ResourceHolder;
|
||||
import io.druid.collections.StupidResourceHolder;
|
||||
import io.druid.guice.GuiceInjectors;
|
||||
import io.druid.guice.ManageLifecycle;
|
||||
import io.druid.initialization.Initialization;
|
||||
|
@ -110,8 +113,12 @@ public class MemcachedCacheTest
|
|||
@Before
|
||||
public void setUp() throws Exception
|
||||
{
|
||||
MemcachedClientIF client = new MockMemcachedClient();
|
||||
cache = new MemcachedCache(client, memcachedCacheConfig);
|
||||
cache = new MemcachedCache(
|
||||
Suppliers.<ResourceHolder<MemcachedClientIF>>ofInstance(
|
||||
StupidResourceHolder.<MemcachedClientIF>create(new MockMemcachedClient())
|
||||
),
|
||||
memcachedCacheConfig
|
||||
);
|
||||
}
|
||||
|
||||
@Test
|
||||
|
|
Loading…
Reference in New Issue