mirror of https://github.com/apache/lucene.git
LUCENE-4025: add RefereneManager.maybeRefreshBlocking()
git-svn-id: https://svn.apache.org/repos/asf/lucene/dev/trunk@1332699 13f79535-47bb-0310-9956-ffa450edef68
This commit is contained in:
parent
5ede77e922
commit
77ababf249
|
@ -846,6 +846,9 @@ New features
|
|||
* LUCENE-4004: Add DisjunctionMaxQuery support to the xml query parser.
|
||||
(Benson Margulies via Robert Muir)
|
||||
|
||||
* LUCENE-4025: Add maybeRefreshBlocking to ReferenceManager, to let a caller
|
||||
block until the refresh logic has been executed. (Shai Erera, Mike McCandless)
|
||||
|
||||
Optimizations
|
||||
|
||||
* LUCENE-2588: Don't store unnecessary suffixes when writing the terms
|
||||
|
|
|
@ -19,7 +19,8 @@ package org.apache.lucene.search;
|
|||
|
||||
import java.io.Closeable;
|
||||
import java.io.IOException;
|
||||
import java.util.concurrent.Semaphore;
|
||||
import java.util.concurrent.locks.Lock;
|
||||
import java.util.concurrent.locks.ReentrantLock;
|
||||
|
||||
import org.apache.lucene.store.AlreadyClosedException;
|
||||
|
||||
|
@ -42,7 +43,7 @@ public abstract class ReferenceManager<G> implements Closeable {
|
|||
|
||||
protected volatile G current;
|
||||
|
||||
private final Semaphore reopenLock = new Semaphore(1);
|
||||
private final Lock refreshLock = new ReentrantLock();
|
||||
|
||||
private void ensureOpen() {
|
||||
if (current == null) {
|
||||
|
@ -108,29 +109,14 @@ public abstract class ReferenceManager<G> implements Closeable {
|
|||
protected void afterClose() throws IOException {
|
||||
}
|
||||
|
||||
/**
|
||||
* You must call this, periodically, if you want that {@link #acquire()} will
|
||||
* return refreshed instances.
|
||||
*
|
||||
* <p>
|
||||
* <b>Threads</b>: it's fine for more than one thread to call this at once.
|
||||
* Only the first thread will attempt the refresh; subsequent threads will see
|
||||
* that another thread is already handling refresh and will return
|
||||
* immediately. Note that this means if another thread is already refreshing
|
||||
* then subsequent threads will return right away without waiting for the
|
||||
* refresh to complete.
|
||||
*
|
||||
* <p>
|
||||
* If this method returns true it means the calling thread either refreshed
|
||||
* or that there were no changes to refresh. If it returns false it means another
|
||||
* thread is currently refreshing.
|
||||
*/
|
||||
public final boolean maybeRefresh() throws IOException {
|
||||
ensureOpen();
|
||||
|
||||
// Ensure only 1 thread does reopen at once; other threads just return immediately:
|
||||
final boolean doTryRefresh = reopenLock.tryAcquire();
|
||||
if (doTryRefresh) {
|
||||
private void doMaybeRefresh() throws IOException {
|
||||
// it's ok to call lock() here (blocking) because we're supposed to get here
|
||||
// from either maybeRefreh() or maybeRefreshBlocking(), after the lock has
|
||||
// already been obtained. Doing that protects us from an accidental bug
|
||||
// where this method will be called outside the scope of refreshLock.
|
||||
// Per ReentrantLock's javadoc, calling lock() by the same thread more than
|
||||
// once is ok, as long as unlock() is called a matching number of times.
|
||||
refreshLock.lock();
|
||||
try {
|
||||
final G reference = acquire();
|
||||
try {
|
||||
|
@ -152,13 +138,66 @@ public abstract class ReferenceManager<G> implements Closeable {
|
|||
}
|
||||
afterRefresh();
|
||||
} finally {
|
||||
reopenLock.release();
|
||||
refreshLock.unlock();
|
||||
}
|
||||
}
|
||||
|
||||
/**
|
||||
* You must call this (or {@link #maybeRefreshBlocking()}), periodically, if
|
||||
* you want that {@link #acquire()} will return refreshed instances.
|
||||
*
|
||||
* <p>
|
||||
* <b>Threads</b>: it's fine for more than one thread to call this at once.
|
||||
* Only the first thread will attempt the refresh; subsequent threads will see
|
||||
* that another thread is already handling refresh and will return
|
||||
* immediately. Note that this means if another thread is already refreshing
|
||||
* then subsequent threads will return right away without waiting for the
|
||||
* refresh to complete.
|
||||
*
|
||||
* <p>
|
||||
* If this method returns true it means the calling thread either refreshed or
|
||||
* that there were no changes to refresh. If it returns false it means another
|
||||
* thread is currently refreshing.
|
||||
*/
|
||||
public final boolean maybeRefresh() throws IOException {
|
||||
ensureOpen();
|
||||
|
||||
// Ensure only 1 thread does reopen at once; other threads just return immediately:
|
||||
final boolean doTryRefresh = refreshLock.tryLock();
|
||||
if (doTryRefresh) {
|
||||
try {
|
||||
doMaybeRefresh();
|
||||
} finally {
|
||||
refreshLock.unlock();
|
||||
}
|
||||
}
|
||||
|
||||
return doTryRefresh;
|
||||
}
|
||||
|
||||
/**
|
||||
* You must call this (or {@link #maybeRefresh()}), periodically, if you want
|
||||
* that {@link #acquire()} will return refreshed instances.
|
||||
*
|
||||
* <p>
|
||||
* <b>Threads</b>: unlike {@link #maybeRefresh()}, if another thread is
|
||||
* currently refreshing, this method blocks until that thread completes. It is
|
||||
* useful if you want to guarantee that the next call to {@link #acquire()}
|
||||
* will return a refreshed instance. Otherwise, consider using the
|
||||
* non-blocking {@link #maybeRefresh()}.
|
||||
*/
|
||||
public final void maybeRefreshBlocking() throws IOException, InterruptedException {
|
||||
ensureOpen();
|
||||
|
||||
// Ensure only 1 thread does reopen at once
|
||||
refreshLock.lock();
|
||||
try {
|
||||
doMaybeRefresh();
|
||||
} finally {
|
||||
refreshLock.lock();
|
||||
}
|
||||
}
|
||||
|
||||
/** Called after swapReference has installed a new
|
||||
* instance. */
|
||||
protected void afterRefresh() throws IOException {
|
||||
|
|
|
@ -108,7 +108,11 @@ public class TestSearcherManager extends ThreadedIndexingAndSearchingTestCase {
|
|||
Thread.sleep(_TestUtil.nextInt(random(), 1, 100));
|
||||
writer.commit();
|
||||
Thread.sleep(_TestUtil.nextInt(random(), 1, 5));
|
||||
if (mgr.maybeRefresh()) {
|
||||
boolean block = random().nextBoolean();
|
||||
if (block) {
|
||||
mgr.maybeRefreshBlocking();
|
||||
lifetimeMGR.prune(pruner);
|
||||
} else if (mgr.maybeRefresh()) {
|
||||
lifetimeMGR.prune(pruner);
|
||||
}
|
||||
}
|
||||
|
|
Loading…
Reference in New Issue