Skip to content

Commit 650f044

Browse files
Kennethkeith-turner
authored andcommitted
fixes #938 made tx info cache configurable (#941)
1 parent c737df6 commit 650f044

File tree

2 files changed

+49
-3
lines changed

2 files changed

+49
-3
lines changed

modules/core/src/main/java/org/apache/fluo/core/impl/FluoConfigurationImpl.java

Lines changed: 42 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -15,6 +15,8 @@
1515

1616
package org.apache.fluo.core.impl;
1717

18+
import java.util.concurrent.TimeUnit;
19+
1820
import org.apache.fluo.api.config.FluoConfiguration;
1921

2022
/**
@@ -102,6 +104,46 @@ public static int getTxCommitMemory(FluoConfiguration conf) {
102104
return m;
103105
}
104106

107+
public static final String TX_INFO_CACHE_SIZE = FLUO_IMPL_PREFIX + ".tx.failed.cache.size.mb";
108+
public static final long TX_INFO_CACHE_SIZE_DEFAULT = 10000000;
109+
110+
/**
111+
* Gets the cache size
112+
*
113+
* @param conf The FluoConfiguration
114+
* @return The size of the cache value from the property value {@value #TX_INFO_CACHE_SIZE}
115+
* if it is set, else the value of the default value {@value #TX_INFO_CACHE_SIZE_DEFAULT}
116+
*/
117+
118+
public static long getTxInfoCacheSize(FluoConfiguration conf) {
119+
long size = conf.getLong(TX_INFO_CACHE_SIZE, TX_INFO_CACHE_SIZE_DEFAULT);
120+
if (size <= 0) {
121+
throw new IllegalArgumentException("Cache size must be positive for " + TX_INFO_CACHE_SIZE);
122+
}
123+
return size;
124+
}
125+
126+
public static final String TX_INFO_CACHE_TIMEOUT =
127+
FLUO_IMPL_PREFIX + ".tx.failed.cache.expireTime.ms";
128+
public static final long TX_INFO_CACHE_TIMEOUT_DEFAULT = 24 * 60 * 1000;
129+
130+
/**
131+
* Gets the time before stale entries in the cache are evicted based on age.
132+
* This method returns a long representing the time converted from the
133+
* TimeUnit passed in.
134+
*
135+
* @param conf The FluoConfiguration
136+
* @param tu The TimeUnit desired to represent the cache timeout
137+
*/
138+
139+
public static long getTxIfoCacheTimeout(FluoConfiguration conf, TimeUnit tu) {
140+
long millis = conf.getLong(TX_INFO_CACHE_TIMEOUT, TX_INFO_CACHE_TIMEOUT_DEFAULT);
141+
if (millis <= 0) {
142+
throw new IllegalArgumentException("Timeout must positive for " + TX_INFO_CACHE_TIMEOUT);
143+
}
144+
return tu.convert(millis, TimeUnit.MILLISECONDS);
145+
}
146+
105147
public static final String ASYNC_CW_THREADS = FLUO_IMPL_PREFIX + ".async.cw.threads";
106148
public static final int ASYNC_CW_THREADS_DEFAULT = 8;
107149
public static final String ASYNC_CW_LIMIT = FLUO_IMPL_PREFIX + ".async.cw.limit";

modules/core/src/main/java/org/apache/fluo/core/impl/TxInfoCache.java

Lines changed: 7 additions & 3 deletions
Original file line numberDiff line numberDiff line change
@@ -20,6 +20,7 @@
2020
import com.google.common.cache.Cache;
2121
import com.google.common.cache.CacheBuilder;
2222
import com.google.common.cache.Weigher;
23+
import org.apache.fluo.api.config.FluoConfiguration;
2324
import org.apache.fluo.api.data.Bytes;
2425
import org.apache.fluo.api.data.Column;
2526

@@ -38,8 +39,12 @@ public int weigh(PrimaryRowColumn key, TxInfo value) {
3839
private final Environment env;
3940

4041
TxInfoCache(Environment env) {
41-
cache = CacheBuilder.newBuilder().expireAfterAccess(CACHE_TIMEOUT_MIN, TimeUnit.MINUTES)
42-
.maximumWeight(10000000).weigher(new TxStatusWeigher()).concurrencyLevel(10).build();
42+
final FluoConfiguration conf = env.getConfiguration();
43+
cache = CacheBuilder.newBuilder()
44+
.expireAfterAccess(FluoConfigurationImpl.getTxIfoCacheTimeout(conf, TimeUnit.MILLISECONDS),
45+
TimeUnit.MILLISECONDS)
46+
.maximumWeight(FluoConfigurationImpl.getTxInfoCacheSize(conf))
47+
.weigher(new TxStatusWeigher()).concurrencyLevel(10).build();
4348
this.env = env;
4449
}
4550

@@ -58,7 +63,6 @@ public TxInfo getTransactionInfo(PrimaryRowColumn key) {
5863
cache.put(key, txInfo);
5964
}
6065
}
61-
6266
return txInfo;
6367
}
6468
}

0 commit comments

Comments
 (0)