SOLR-8750: replace anonymous inner class for callable, Runnable etc

This commit is contained in:
Noble Paul 2016-04-05 22:45:36 +05:30
parent 784e3e3863
commit d67ec54932
26 changed files with 210 additions and 370 deletions

View File

@ -28,15 +28,11 @@ public abstract class Expression {
public abstract Comparable getValue(); public abstract Comparable getValue();
public Comparator<Expression> comparator(final FacetSortDirection direction) { public Comparator<Expression> comparator(final FacetSortDirection direction) {
return new Comparator<Expression>(){ return (a, b) -> {
@SuppressWarnings("unchecked") if( direction == FacetSortDirection.ASCENDING ){
@Override return a.getValue().compareTo(b.getValue());
public int compare(Expression a, Expression b) { } else {
if( direction == FacetSortDirection.ASCENDING ){ return b.getValue().compareTo(a.getValue());
return a.getValue().compareTo(b.getValue());
} else {
return b.getValue().compareTo(a.getValue());
}
} }
}; };
} }

View File

@ -87,22 +87,19 @@ class GoLive {
baseUrl = baseUrl.substring(0, lastPathIndex); baseUrl = baseUrl.substring(0, lastPathIndex);
final String mergeUrl = baseUrl; final String mergeUrl = baseUrl;
Callable<Request> task = new Callable<Request>() { Callable<Request> task = () -> {
@Override Request req = new Request();
public Request call() { LOG.info("Live merge " + dir.getPath() + " into " + mergeUrl);
Request req = new Request(); try (final HttpSolrClient client = new HttpSolrClient(mergeUrl)) {
LOG.info("Live merge " + dir.getPath() + " into " + mergeUrl); CoreAdminRequest.MergeIndexes mergeRequest = new CoreAdminRequest.MergeIndexes();
try (final HttpSolrClient client = new HttpSolrClient(mergeUrl)) { mergeRequest.setCoreName(name);
CoreAdminRequest.MergeIndexes mergeRequest = new CoreAdminRequest.MergeIndexes(); mergeRequest.setIndexDirs(Arrays.asList(dir.getPath().toString() + "/data/index"));
mergeRequest.setCoreName(name); mergeRequest.process(client);
mergeRequest.setIndexDirs(Arrays.asList(dir.getPath().toString() + "/data/index")); req.success = true;
mergeRequest.process(client); } catch (SolrServerException | IOException e) {
req.success = true; req.e = e;
} catch (SolrServerException | IOException e) {
req.e = e;
}
return req;
} }
return req;
}; };
pending.add(completionService.submit(task)); pending.add(completionService.submit(task));
} }

View File

@ -1188,12 +1188,7 @@ public class MapReduceIndexerTool extends Configured implements Tool {
} }
// use alphanumeric sort (rather than lexicographical sort) to properly handle more than 99999 shards // use alphanumeric sort (rather than lexicographical sort) to properly handle more than 99999 shards
Arrays.sort(dirs, new Comparator<FileStatus>() { Arrays.sort(dirs, (f1, f2) -> new AlphaNumericComparator().compare(f1.getPath().getName(), f2.getPath().getName()));
@Override
public int compare(FileStatus f1, FileStatus f2) {
return new AlphaNumericComparator().compare(f1.getPath().getName(), f2.getPath().getName());
}
});
return dirs; return dirs;
} }

View File

@ -114,12 +114,9 @@ final class ZooKeeperInspector {
public List<Slice> getSortedSlices(Collection<Slice> slices) { public List<Slice> getSortedSlices(Collection<Slice> slices) {
List<Slice> sorted = new ArrayList(slices); List<Slice> sorted = new ArrayList(slices);
Collections.sort(sorted, new Comparator<Slice>() { Collections.sort(sorted, (slice1, slice2) -> {
@Override Comparator c = new AlphaNumericComparator();
public int compare(Slice slice1, Slice slice2) { return c.compare(slice1.getName(), slice2.getName());
Comparator c = new AlphaNumericComparator();
return c.compare(slice1.getName(), slice2.getName());
}
}); });
LOG.trace("Sorted slices: {}", sorted); LOG.trace("Sorted slices: {}", sorted);
return sorted; return sorted;

View File

@ -79,12 +79,7 @@ public class SolrMorphlineZkAvroTest extends AbstractSolrMorphlineZkTestBase {
assertEquals(2104, collector.getRecords().size()); assertEquals(2104, collector.getRecords().size());
assertEquals(collector.getRecords().size(), rsp.getResults().size()); assertEquals(collector.getRecords().size(), rsp.getResults().size());
Collections.sort(collector.getRecords(), new Comparator<Record>() { Collections.sort(collector.getRecords(), (r1, r2) -> r1.get("id").toString().compareTo(r2.get("id").toString()));
@Override
public int compare(Record r1, Record r2) {
return r1.get("id").toString().compareTo(r2.get("id").toString());
}
});
// fetch test input data and sort like solr result set // fetch test input data and sort like solr result set
List<GenericData.Record> records = new ArrayList(); List<GenericData.Record> records = new ArrayList();
@ -94,12 +89,7 @@ public class SolrMorphlineZkAvroTest extends AbstractSolrMorphlineZkTestBase {
records.add(expected); records.add(expected);
} }
assertEquals(collector.getRecords().size(), records.size()); assertEquals(collector.getRecords().size(), records.size());
Collections.sort(records, new Comparator<GenericData.Record>() { Collections.sort(records, (r1, r2) -> r1.get("id").toString().compareTo(r2.get("id").toString()));
@Override
public int compare(GenericData.Record r1, GenericData.Record r2) {
return r1.get("id").toString().compareTo(r2.get("id").toString());
}
});
Object lastId = null; Object lastId = null;
for (int i = 0; i < records.size(); i++) { for (int i = 0; i < records.size(); i++) {

View File

@ -248,13 +248,7 @@ public class OverseerAutoReplicaFailoverThread implements Runnable, Closeable {
log.debug("submit call to {}", createUrl); log.debug("submit call to {}", createUrl);
MDC.put("OverseerAutoReplicaFailoverThread.createUrl", createUrl); MDC.put("OverseerAutoReplicaFailoverThread.createUrl", createUrl);
try { try {
updateExecutor.submit(new Callable<Boolean>() { updateExecutor.submit(() -> createSolrCore(collection, createUrl, dataDir, ulogDir, coreNodeName, coreName));
@Override
public Boolean call() {
return createSolrCore(collection, createUrl, dataDir, ulogDir, coreNodeName, coreName);
}
});
} finally { } finally {
MDC.remove("OverseerAutoReplicaFailoverThread.createUrl"); MDC.remove("OverseerAutoReplicaFailoverThread.createUrl");
} }

View File

@ -204,12 +204,9 @@ public abstract class ConfigSetService {
if (Files.exists(schemaFile)) { if (Files.exists(schemaFile)) {
try { try {
String cachedName = cacheName(schemaFile); String cachedName = cacheName(schemaFile);
return schemaCache.get(cachedName, new Callable<IndexSchema>() { return schemaCache.get(cachedName, () -> {
@Override logger.info("Creating new index schema for core {}", cd.getName());
public IndexSchema call() throws Exception { return IndexSchemaFactory.buildIndexSchema(cd.getSchemaName(), solrConfig);
logger.info("Creating new index schema for core {}", cd.getName());
return IndexSchemaFactory.buildIndexSchema(cd.getSchemaName(), solrConfig);
}
}); });
} catch (ExecutionException e) { } catch (ExecutionException e) {
throw new SolrException(SolrException.ErrorCode.SERVER_ERROR, throw new SolrException(SolrException.ErrorCode.SERVER_ERROR,

View File

@ -456,28 +456,25 @@ public class CoreContainer {
solrCores.markCoreAsLoading(cd); solrCores.markCoreAsLoading(cd);
} }
if (cd.isLoadOnStartup()) { if (cd.isLoadOnStartup()) {
futures.add(coreLoadExecutor.submit(new Callable<SolrCore>() { futures.add(coreLoadExecutor.submit(() -> {
@Override SolrCore core;
public SolrCore call() throws Exception { try {
SolrCore core; if (zkSys.getZkController() != null) {
try { zkSys.getZkController().throwErrorIfReplicaReplaced(cd);
if (zkSys.getZkController() != null) { }
zkSys.getZkController().throwErrorIfReplicaReplaced(cd);
}
core = create(cd, false); core = create(cd, false);
} finally { } finally {
if (asyncSolrCoreLoad) { if (asyncSolrCoreLoad) {
solrCores.markCoreAsNotLoading(cd); solrCores.markCoreAsNotLoading(cd);
}
} }
try {
zkSys.registerInZk(core, true);
} catch (RuntimeException e) {
SolrException.log(log, "Error registering SolrCore", e);
}
return core;
} }
try {
zkSys.registerInZk(core, true);
} catch (RuntimeException e) {
SolrException.log(log, "Error registering SolrCore", e);
}
return core;
})); }));
} }
} }

View File

@ -318,12 +318,7 @@ public class HdfsDirectoryFactory extends CachingDirectoryFactory implements Sol
FileSystem fileSystem = null; FileSystem fileSystem = null;
try { try {
// no need to close the fs, the cache will do it // no need to close the fs, the cache will do it
fileSystem = tmpFsCache.get(path, new Callable<FileSystem>() { fileSystem = tmpFsCache.get(path, () -> FileSystem.get(hdfsDirPath.toUri(), conf));
@Override
public FileSystem call() throws IOException {
return FileSystem.get(hdfsDirPath.toUri(), conf);
}
});
} catch (ExecutionException e) { } catch (ExecutionException e) {
throw new RuntimeException(e); throw new RuntimeException(e);
} }
@ -351,12 +346,7 @@ public class HdfsDirectoryFactory extends CachingDirectoryFactory implements Sol
try { try {
// no need to close the fs, the cache will do it // no need to close the fs, the cache will do it
fileSystem = tmpFsCache.get(cacheValue.path, new Callable<FileSystem>() { fileSystem = tmpFsCache.get(cacheValue.path, () -> FileSystem.get(new Path(cacheValue.path).toUri(), conf));
@Override
public FileSystem call() throws IOException {
return FileSystem.get(new Path(cacheValue.path).toUri(), conf);
}
});
} catch (ExecutionException e) { } catch (ExecutionException e) {
throw new RuntimeException(e); throw new RuntimeException(e);
} }
@ -487,12 +477,7 @@ public class HdfsDirectoryFactory extends CachingDirectoryFactory implements Sol
final Configuration conf = getConf(); final Configuration conf = getConf();
FileSystem fileSystem = null; FileSystem fileSystem = null;
try { try {
fileSystem = tmpFsCache.get(dataDir, new Callable<FileSystem>() { fileSystem = tmpFsCache.get(dataDir, () -> FileSystem.get(dataDirPath.toUri(), conf));
@Override
public FileSystem call() throws IOException {
return FileSystem.get(dataDirPath.toUri(), conf);
}
});
} catch (ExecutionException e) { } catch (ExecutionException e) {
throw new RuntimeException(e); throw new RuntimeException(e);
} }

View File

@ -735,12 +735,9 @@ public final class SolrCore implements SolrInfoMBean, Closeable {
// cause the executor to stall so firstSearcher events won't fire // cause the executor to stall so firstSearcher events won't fire
// until after inform() has been called for all components. // until after inform() has been called for all components.
// searchExecutor must be single-threaded for this to work // searchExecutor must be single-threaded for this to work
searcherExecutor.submit(new Callable<Void>() { searcherExecutor.submit(() -> {
@Override latch.await();
public Void call() throws Exception { return null;
latch.await();
return null;
}
}); });
this.updateHandler = initUpdateHandler(updateHandler); this.updateHandler = initUpdateHandler(updateHandler);
@ -854,14 +851,7 @@ public final class SolrCore implements SolrInfoMBean, Closeable {
if (iwRef != null) { if (iwRef != null) {
final IndexWriter iw = iwRef.get(); final IndexWriter iw = iwRef.get();
final SolrCore core = this; final SolrCore core = this;
newReaderCreator = new Callable<DirectoryReader>() { newReaderCreator = () -> indexReaderFactory.newReader(iw, core);
// this is used during a core reload
@Override
public DirectoryReader call() throws Exception {
return indexReaderFactory.newReader(iw, core);
}
};
} }
} }
@ -1779,57 +1769,48 @@ public final class SolrCore implements SolrInfoMBean, Closeable {
// warm the new searcher based on the current searcher. // warm the new searcher based on the current searcher.
// should this go before the other event handlers or after? // should this go before the other event handlers or after?
if (currSearcher != null) { if (currSearcher != null) {
future = searcherExecutor.submit(new Callable() { future = searcherExecutor.submit(() -> {
@Override try {
public Object call() throws Exception { newSearcher.warm(currSearcher);
try { } catch (Throwable e) {
newSearcher.warm(currSearcher); SolrException.log(log, e);
} catch (Throwable e) { if (e instanceof Error) {
SolrException.log(log, e); throw (Error) e;
if (e instanceof Error) {
throw (Error) e;
}
} }
return null;
} }
return null;
}); });
} }
if (currSearcher == null) { if (currSearcher == null) {
future = searcherExecutor.submit(new Callable() { future = searcherExecutor.submit(() -> {
@Override try {
public Object call() throws Exception { for (SolrEventListener listener : firstSearcherListeners) {
try { listener.newSearcher(newSearcher, null);
for (SolrEventListener listener : firstSearcherListeners) { }
listener.newSearcher(newSearcher, null); } catch (Throwable e) {
} SolrException.log(log, null, e);
} catch (Throwable e) { if (e instanceof Error) {
SolrException.log(log, null, e); throw (Error) e;
if (e instanceof Error) {
throw (Error) e;
}
} }
return null;
} }
return null;
}); });
} }
if (currSearcher != null) { if (currSearcher != null) {
future = searcherExecutor.submit(new Callable() { future = searcherExecutor.submit(() -> {
@Override try {
public Object call() throws Exception { for (SolrEventListener listener : newSearcherListeners) {
try { listener.newSearcher(newSearcher, currSearcher);
for (SolrEventListener listener : newSearcherListeners) { }
listener.newSearcher(newSearcher, currSearcher); } catch (Throwable e) {
} SolrException.log(log, null, e);
} catch (Throwable e) { if (e instanceof Error) {
SolrException.log(log, null, e); throw (Error) e;
if (e instanceof Error) {
throw (Error) e;
}
} }
return null;
} }
return null;
}); });
} }

View File

@ -127,22 +127,19 @@ class SolrCores {
new DefaultSolrThreadFactory("coreCloseExecutor")); new DefaultSolrThreadFactory("coreCloseExecutor"));
try { try {
for (SolrCore core : coreList) { for (SolrCore core : coreList) {
coreCloseExecutor.submit(new Callable<SolrCore>() { coreCloseExecutor.submit(() -> {
@Override MDCLoggingContext.setCore(core);
public SolrCore call() throws Exception { try {
MDCLoggingContext.setCore(core); core.close();
try { } catch (Throwable e) {
core.close(); SolrException.log(log, "Error shutting down core", e);
} catch (Throwable e) { if (e instanceof Error) {
SolrException.log(log, "Error shutting down core", e); throw (Error) e;
if (e instanceof Error) {
throw (Error) e;
}
} finally {
MDCLoggingContext.clear();
} }
return core; } finally {
MDCLoggingContext.clear();
} }
return core;
}); });
} }
} finally { } finally {

View File

@ -73,19 +73,14 @@ class CdcrReplicatorScheduler {
for (int i = 0; i < nCandidates; i++) { for (int i = 0; i < nCandidates; i++) {
// a thread that poll one state from the queue, execute the replication task, and push back // a thread that poll one state from the queue, execute the replication task, and push back
// the state in the queue when the task is completed // the state in the queue when the task is completed
replicatorsPool.execute(new Runnable() { replicatorsPool.execute(() -> {
CdcrReplicatorState state = statesQueue.poll();
@Override assert state != null; // Should never happen
public void run() { try {
CdcrReplicatorState state = statesQueue.poll(); new CdcrReplicator(state, batchSize).run();
assert state != null; // Should never happen } finally {
try { statesQueue.offer(state);
new CdcrReplicator(state, batchSize).run();
} finally {
statesQueue.offer(state);
}
} }
}); });
} }

View File

@ -284,17 +284,7 @@ public class MoreLikeThisHandler extends RequestHandlerBase
{ {
public Term term; public Term term;
public float boost; public float boost;
public static Comparator<InterestingTerm> BOOST_ORDER = new Comparator<InterestingTerm>() {
@Override
public int compare(InterestingTerm t1, InterestingTerm t2) {
float d = t1.boost - t2.boost;
if( d == 0 ) {
return 0;
}
return (d>0)?1:-1;
}
};
} }
/** /**

View File

@ -160,64 +160,61 @@ public class HttpShardHandler extends ShardHandler {
// do this outside of the callable for thread safety reasons // do this outside of the callable for thread safety reasons
final List<String> urls = getURLs(shard, preferredHostAddress); final List<String> urls = getURLs(shard, preferredHostAddress);
Callable<ShardResponse> task = new Callable<ShardResponse>() { Callable<ShardResponse> task = () -> {
@Override
public ShardResponse call() throws Exception {
ShardResponse srsp = new ShardResponse(); ShardResponse srsp = new ShardResponse();
if (sreq.nodeName != null) { if (sreq.nodeName != null) {
srsp.setNodeName(sreq.nodeName); srsp.setNodeName(sreq.nodeName);
}
srsp.setShardRequest(sreq);
srsp.setShard(shard);
SimpleSolrResponse ssr = new SimpleSolrResponse();
srsp.setSolrResponse(ssr);
long startTime = System.nanoTime();
try {
params.remove(CommonParams.WT); // use default (currently javabin)
params.remove(CommonParams.VERSION);
QueryRequest req = makeQueryRequest(sreq, params, shard);
req.setMethod(SolrRequest.METHOD.POST);
// no need to set the response parser as binary is the default
// req.setResponseParser(new BinaryResponseParser());
// if there are no shards available for a slice, urls.size()==0
if (urls.size()==0) {
// TODO: what's the right error code here? We should use the same thing when
// all of the servers for a shard are down.
throw new SolrException(SolrException.ErrorCode.SERVICE_UNAVAILABLE, "no servers hosting shard: " + shard);
}
if (urls.size() <= 1) {
String url = urls.get(0);
srsp.setShardAddress(url);
try (SolrClient client = new HttpSolrClient(url, httpClient)) {
ssr.nl = client.request(req);
}
} else {
LBHttpSolrClient.Rsp rsp = httpShardHandlerFactory.makeLoadBalancedRequest(req, urls);
ssr.nl = rsp.getResponse();
srsp.setShardAddress(rsp.getServer());
}
}
catch( ConnectException cex ) {
srsp.setException(cex); //????
} catch (Exception th) {
srsp.setException(th);
if (th instanceof SolrException) {
srsp.setResponseCode(((SolrException)th).code());
} else {
srsp.setResponseCode(-1);
}
}
ssr.elapsedTime = TimeUnit.MILLISECONDS.convert(System.nanoTime() - startTime, TimeUnit.NANOSECONDS);
return transfomResponse(sreq, srsp, shard);
} }
srsp.setShardRequest(sreq);
srsp.setShard(shard);
SimpleSolrResponse ssr = new SimpleSolrResponse();
srsp.setSolrResponse(ssr);
long startTime = System.nanoTime();
try {
params.remove(CommonParams.WT); // use default (currently javabin)
params.remove(CommonParams.VERSION);
QueryRequest req = makeQueryRequest(sreq, params, shard);
req.setMethod(SolrRequest.METHOD.POST);
// no need to set the response parser as binary is the default
// req.setResponseParser(new BinaryResponseParser());
// if there are no shards available for a slice, urls.size()==0
if (urls.size()==0) {
// TODO: what's the right error code here? We should use the same thing when
// all of the servers for a shard are down.
throw new SolrException(SolrException.ErrorCode.SERVICE_UNAVAILABLE, "no servers hosting shard: " + shard);
}
if (urls.size() <= 1) {
String url = urls.get(0);
srsp.setShardAddress(url);
try (SolrClient client = new HttpSolrClient(url, httpClient)) {
ssr.nl = client.request(req);
}
} else {
LBHttpSolrClient.Rsp rsp = httpShardHandlerFactory.makeLoadBalancedRequest(req, urls);
ssr.nl = rsp.getResponse();
srsp.setShardAddress(rsp.getServer());
}
}
catch( ConnectException cex ) {
srsp.setException(cex); //????
} catch (Exception th) {
srsp.setException(th);
if (th instanceof SolrException) {
srsp.setResponseCode(((SolrException)th).code());
} else {
srsp.setResponseCode(-1);
}
}
ssr.elapsedTime = TimeUnit.MILLISECONDS.convert(System.nanoTime() - startTime, TimeUnit.NANOSECONDS);
return transfomResponse(sreq, srsp, shard);
}; };
try { try {

View File

@ -103,12 +103,9 @@ class PerSegmentSingleValuedFaceting {
for (final LeafReaderContext leave : leaves) { for (final LeafReaderContext leave : leaves) {
final SegFacet segFacet = new SegFacet(leave); final SegFacet segFacet = new SegFacet(leave);
Callable<SegFacet> task = new Callable<SegFacet>() { Callable<SegFacet> task = () -> {
@Override segFacet.countTerms();
public SegFacet call() throws Exception { return segFacet;
segFacet.countTerms();
return segFacet;
}
}; };
// TODO: if limiting threads, submit by largest segment first? // TODO: if limiting threads, submit by largest segment first?

View File

@ -662,26 +662,23 @@ public class SimpleFacets {
final String termList = localParams == null ? null : localParams.get(CommonParams.TERMS); final String termList = localParams == null ? null : localParams.get(CommonParams.TERMS);
final String key = parsed.key; final String key = parsed.key;
final String facetValue = parsed.facetValue; final String facetValue = parsed.facetValue;
Callable<NamedList> callable = new Callable<NamedList>() { Callable<NamedList> callable = () -> {
@Override try {
public NamedList call() throws Exception { NamedList<Object> result = new SimpleOrderedMap<>();
try { if(termList != null) {
NamedList<Object> result = new SimpleOrderedMap<>(); List<String> terms = StrUtils.splitSmart(termList, ",", true);
if(termList != null) { result.add(key, getListedTermCounts(facetValue, parsed, terms));
List<String> terms = StrUtils.splitSmart(termList, ",", true); } else {
result.add(key, getListedTermCounts(facetValue, parsed, terms)); result.add(key, getTermCounts(facetValue, parsed));
} else {
result.add(key, getTermCounts(facetValue, parsed));
}
return result;
} catch (SolrException se) {
throw se;
} catch (Exception e) {
throw new SolrException(ErrorCode.SERVER_ERROR,
"Exception during facet.field: " + facetValue, e);
} finally {
semaphore.release();
} }
return result;
} catch (SolrException se) {
throw se;
} catch (Exception e) {
throw new SolrException(ErrorCode.SERVER_ERROR,
"Exception during facet.field: " + facetValue, e);
} finally {
semaphore.release();
} }
}; };

View File

@ -407,12 +407,7 @@ public abstract class AbstractSpatialFieldType<T extends SpatialStrategy> extend
*/ */
public T getStrategy(final String fieldName) { public T getStrategy(final String fieldName) {
try { try {
return fieldStrategyCache.get(fieldName, new Callable<T>() { return fieldStrategyCache.get(fieldName, () -> newSpatialStrategy(fieldName));
@Override
public T call() throws Exception {
return newSpatialStrategy(fieldName);
}
});
} catch (ExecutionException e) { } catch (ExecutionException e) {
throw Throwables.propagate(e.getCause()); throw Throwables.propagate(e.getCause());
} }

View File

@ -279,14 +279,9 @@ public class SolrCmdDistributor {
// a commit using ConncurrentUpdateSolrServer is not async, // a commit using ConncurrentUpdateSolrServer is not async,
// so we make it async to prevent commits from happening // so we make it async to prevent commits from happening
// serially across multiple nodes // serially across multiple nodes
pending.add(completionService.submit(new Callable<Object>() { pending.add(completionService.submit(() -> {
doRequest(req);
@Override return null;
public Object call() throws Exception {
doRequest(req);
return null;
}
})); }));
} else { } else {
doRequest(req); doRequest(req);

View File

@ -197,12 +197,7 @@ public class TestRecovery extends SolrTestCaseJ4 {
} }
}; };
UpdateLog.testing_logReplayFinishHook = new Runnable() { UpdateLog.testing_logReplayFinishHook = logReplayFinish::release;
@Override
public void run() {
logReplayFinish.release();
}
};
SolrQueryRequest req = req(); SolrQueryRequest req = req();

View File

@ -161,7 +161,7 @@ public class TestRecoveryHdfs extends SolrTestCaseJ4 {
} }
}; };
UpdateLog.testing_logReplayFinishHook = () -> logReplayFinish.release(); UpdateLog.testing_logReplayFinishHook = logReplayFinish::release;
clearIndex(); clearIndex();
@ -257,7 +257,7 @@ public class TestRecoveryHdfs extends SolrTestCaseJ4 {
} }
}; };
UpdateLog.testing_logReplayFinishHook = () -> logReplayFinish.release(); UpdateLog.testing_logReplayFinishHook = logReplayFinish::release;
SolrQueryRequest req = req(); SolrQueryRequest req = req();
@ -412,7 +412,7 @@ public class TestRecoveryHdfs extends SolrTestCaseJ4 {
} }
}; };
UpdateLog.testing_logReplayFinishHook = () -> logReplayFinish.release(); UpdateLog.testing_logReplayFinishHook = logReplayFinish::release;
SolrQueryRequest req = req(); SolrQueryRequest req = req();
@ -722,12 +722,7 @@ public class TestRecoveryHdfs extends SolrTestCaseJ4 {
} }
}; };
UpdateLog.testing_logReplayFinishHook = new Runnable() { UpdateLog.testing_logReplayFinishHook = logReplayFinish::release;
@Override
public void run() {
logReplayFinish.release();
}
};
clearIndex(); clearIndex();

View File

@ -575,22 +575,14 @@ public class AddBlockUpdateTest extends SolrTestCaseJ4 {
for (Document block : blocks) { for (Document block : blocks) {
final String msg = block.asXML(); final String msg = block.asXML();
if (msg.length() > 0) { if (msg.length() > 0) {
rez.add(new Callable<Void>() { rez.add(() -> {
@Override assertBlockU(msg);
public Void call() { return null;
assertBlockU(msg);
return null;
}
}); });
if (rarely()) { if (rarely()) {
rez.add(new Callable<Void>() { rez.add(() -> {
@Override assertBlockU(commit());
public Void call() { return null;
assertBlockU(commit());
return null;
}
}); });
} }
} }

View File

@ -37,20 +37,12 @@ public class TestUpdate extends SolrTestCaseJ4 {
// Test both by running the same test with and without commits // Test both by running the same test with and without commits
// do without commits // do without commits
doUpdateTest(new Callable() { doUpdateTest(() -> null);
@Override
public Object call() throws Exception {
return null;
}
});
// do with commits // do with commits
doUpdateTest(new Callable() { doUpdateTest(() -> {
@Override assertU(commit("softCommit","false"));
public Object call() throws Exception { return null;
assertU(commit("softCommit","false"));
return null;
}
}); });

View File

@ -603,12 +603,7 @@ public class CloudSolrClient extends SolrClient {
final LBHttpSolrClient.Req lbRequest = entry.getValue(); final LBHttpSolrClient.Req lbRequest = entry.getValue();
try { try {
MDC.put("CloudSolrClient.url", url); MDC.put("CloudSolrClient.url", url);
responseFutures.put(url, threadPool.submit(new Callable<NamedList<?>>() { responseFutures.put(url, threadPool.submit(() -> lbClient.request(lbRequest).getResponse()));
@Override
public NamedList<?> call() throws Exception {
return lbClient.request(lbRequest).getResponse();
}
}));
} finally { } finally {
MDC.remove("CloudSolrClient.url"); MDC.remove("CloudSolrClient.url");
} }

View File

@ -278,12 +278,7 @@ public class HttpSolrClient extends SolrClient {
ExecutorService pool = ExecutorUtil.newMDCAwareFixedThreadPool(1, new SolrjNamedThreadFactory("httpUriRequest")); ExecutorService pool = ExecutorUtil.newMDCAwareFixedThreadPool(1, new SolrjNamedThreadFactory("httpUriRequest"));
try { try {
MDC.put("HttpSolrClient.url", baseUrl); MDC.put("HttpSolrClient.url", baseUrl);
mrr.future = pool.submit(new Callable<NamedList<Object>>(){ mrr.future = pool.submit(() -> executeMethod(method, processor));
@Override
public NamedList<Object> call() throws Exception {
return executeMethod(method, processor);
}});
} finally { } finally {
pool.shutdown(); pool.shutdown();

View File

@ -97,34 +97,28 @@ public class FieldComparator implements StreamComparator {
*/ */
private void assignComparator(){ private void assignComparator(){
if(ComparatorOrder.DESCENDING == order){ if(ComparatorOrder.DESCENDING == order){
comparator = new ComparatorLambda() { comparator = (leftTuple, rightTuple) -> {
@Override Comparable leftComp = (Comparable)leftTuple.get(leftFieldName);
public int compare(Tuple leftTuple, Tuple rightTuple) { Comparable rightComp = (Comparable)rightTuple.get(rightFieldName);
Comparable leftComp = (Comparable)leftTuple.get(leftFieldName);
Comparable rightComp = (Comparable)rightTuple.get(rightFieldName); if(leftComp == rightComp){ return 0; } // if both null then they are equal. if both are same ref then are equal
if(null == leftComp){ return 1; }
if(leftComp == rightComp){ return 0; } // if both null then they are equal. if both are same ref then are equal if(null == rightComp){ return -1; }
if(null == leftComp){ return 1; }
if(null == rightComp){ return -1; } return rightComp.compareTo(leftComp);
return rightComp.compareTo(leftComp);
}
}; };
} }
else{ else{
// See above for black magic reasoning. // See above for black magic reasoning.
comparator = new ComparatorLambda() { comparator = (leftTuple, rightTuple) -> {
@Override Comparable leftComp = (Comparable)leftTuple.get(leftFieldName);
public int compare(Tuple leftTuple, Tuple rightTuple) { Comparable rightComp = (Comparable)rightTuple.get(rightFieldName);
Comparable leftComp = (Comparable)leftTuple.get(leftFieldName);
Comparable rightComp = (Comparable)rightTuple.get(rightFieldName); if(leftComp == rightComp){ return 0; } // if both null then they are equal. if both are same ref then are equal
if(null == leftComp){ return -1; }
if(leftComp == rightComp){ return 0; } // if both null then they are equal. if both are same ref then are equal if(null == rightComp){ return 1; }
if(null == leftComp){ return -1; }
if(null == rightComp){ return 1; } return leftComp.compareTo(rightComp);
return leftComp.compareTo(rightComp);
}
}; };
} }
} }

View File

@ -206,12 +206,7 @@ public class MiniSolrCloudCluster {
List<Callable<JettySolrRunner>> startups = new ArrayList<>(numServers); List<Callable<JettySolrRunner>> startups = new ArrayList<>(numServers);
for (int i = 0; i < numServers; ++i) { for (int i = 0; i < numServers; ++i) {
startups.add(new Callable<JettySolrRunner>() { startups.add(() -> startJettySolrRunner(newNodeName(), jettyConfig.context, jettyConfig));
@Override
public JettySolrRunner call() throws Exception {
return startJettySolrRunner(newNodeName(), jettyConfig.context, jettyConfig);
}
});
} }
Collection<Future<JettySolrRunner>> futures = executor.invokeAll(startups); Collection<Future<JettySolrRunner>> futures = executor.invokeAll(startups);
@ -424,12 +419,7 @@ public class MiniSolrCloudCluster {
solrClient.close(); solrClient.close();
List<Callable<JettySolrRunner>> shutdowns = new ArrayList<>(jettys.size()); List<Callable<JettySolrRunner>> shutdowns = new ArrayList<>(jettys.size());
for (final JettySolrRunner jetty : jettys) { for (final JettySolrRunner jetty : jettys) {
shutdowns.add(new Callable<JettySolrRunner>() { shutdowns.add(() -> stopJettySolrRunner(jetty));
@Override
public JettySolrRunner call() throws Exception {
return stopJettySolrRunner(jetty);
}
});
} }
jettys.clear(); jettys.clear();
Collection<Future<JettySolrRunner>> futures = executor.invokeAll(shutdowns); Collection<Future<JettySolrRunner>> futures = executor.invokeAll(shutdowns);