mirror of
https://github.com/apache/lucene.git
synced 2025-03-06 08:19:23 +00:00
patch_29_06_06.diff from LUCENE-620
git-svn-id: https://svn.apache.org/repos/asf/lucene/java/trunk@418363 13f79535-47bb-0310-9956-ffa450edef68
This commit is contained in:
parent
b2dd60bd4b
commit
d13def5f21
@ -18,7 +18,6 @@ package org.apache.lucene.gdata.storage.lucenestorage;
|
||||
|
||||
import java.io.IOException;
|
||||
import java.util.BitSet;
|
||||
import java.util.List;
|
||||
|
||||
import org.apache.lucene.index.IndexReader;
|
||||
import org.apache.lucene.index.Term;
|
||||
@ -42,14 +41,14 @@ public class ModifiedEntryFilter extends Filter {
|
||||
*/
|
||||
private static final long serialVersionUID = -1551686287704213591L;
|
||||
|
||||
private final List<String> entyIds;
|
||||
private final String[] entyIds;
|
||||
|
||||
/**
|
||||
* Creates a new {@link ModifiedEntryFilter}
|
||||
* @param entryIds the entry id's to filter
|
||||
*
|
||||
*/
|
||||
public ModifiedEntryFilter(List<String> entryIds) {
|
||||
public ModifiedEntryFilter(final String[] entryIds) {
|
||||
super();
|
||||
this.entyIds = entryIds;
|
||||
}
|
||||
|
@ -50,7 +50,8 @@ import com.google.gdata.data.Link;
|
||||
* <p>
|
||||
* This implementation uses {@link java.util.concurrent.locks.ReadWriteLock}.
|
||||
* The read lock may be held simultaneously by multiple reader threads, so long
|
||||
* as there are no writers. The write lock is exclusive.</p>
|
||||
* as there are no writers. The write lock is exclusive.
|
||||
* </p>
|
||||
*
|
||||
* @see java.util.concurrent.locks.ReentrantReadWriteLock
|
||||
* @see org.apache.lucene.gdata.storage.lucenestorage.StorageModifier
|
||||
@ -128,7 +129,7 @@ public class StorageBuffer {
|
||||
this.bufferMap.put(feedId, newFeedMap);
|
||||
|
||||
}
|
||||
addLastModified(wrapper.getFeedId(),wrapper.getTimestamp());
|
||||
addLastModified(wrapper.getFeedId(), wrapper.getTimestamp());
|
||||
} finally {
|
||||
/*
|
||||
* add all to exclude from searches doc will be available via the
|
||||
@ -139,22 +140,21 @@ public class StorageBuffer {
|
||||
}
|
||||
}
|
||||
|
||||
private void addLastModified(final String feedId,Long timestamp){
|
||||
if(this.modifiyMap.containsKey(feedId))
|
||||
private void addLastModified(final String feedId, Long timestamp) {
|
||||
if (this.modifiyMap.containsKey(feedId))
|
||||
this.modifiyMap.remove(feedId);
|
||||
this.modifiyMap.put(feedId,timestamp);
|
||||
this.modifiyMap.put(feedId, timestamp);
|
||||
|
||||
}
|
||||
|
||||
protected Long getFeedLastModified(final String feedId){
|
||||
protected Long getFeedLastModified(final String feedId) {
|
||||
return this.modifiyMap.get(feedId);
|
||||
}
|
||||
protected Set<Entry<String,Long>> getLastModified(){
|
||||
|
||||
protected Set<Entry<String, Long>> getLastModified() {
|
||||
return this.modifiyMap.entrySet();
|
||||
}
|
||||
|
||||
|
||||
|
||||
/**
|
||||
* Returns all entries for the given feed id sorted by the update timestamp
|
||||
* desc.
|
||||
@ -203,7 +203,7 @@ public class StorageBuffer {
|
||||
if (tempMap == null)
|
||||
return;
|
||||
tempMap.remove(entryId);
|
||||
this.addLastModified(feedId,new Long(System.currentTimeMillis()));
|
||||
this.addLastModified(feedId, new Long(System.currentTimeMillis()));
|
||||
} finally {
|
||||
this.writeLock.unlock();
|
||||
|
||||
@ -241,17 +241,21 @@ public class StorageBuffer {
|
||||
* available in the lucene index but should not be found during search.
|
||||
*
|
||||
* <p>
|
||||
* this list contains all entries should not be found by the index searcher
|
||||
* This list contains all entries should not be found by the index searcher.
|
||||
* This method creates a copy of the current list to prevent concurrent
|
||||
* modification exceptions while iteration over the collection.
|
||||
* </p>
|
||||
*
|
||||
*
|
||||
* @see ModifiedEntryFilter
|
||||
* @return - a {@link List} of entries to be omitted from a lucene index
|
||||
* @return - a String array of entries to be omitted from a lucene index
|
||||
* search
|
||||
*/
|
||||
public List<String> getExculdList() {
|
||||
public String[] getExculdList() {
|
||||
this.readLock.lock();
|
||||
try {
|
||||
return this.excludeList;
|
||||
return this.excludeList
|
||||
.toArray(new String[this.excludeList.size()]);
|
||||
} finally {
|
||||
this.readLock.unlock();
|
||||
}
|
||||
@ -278,8 +282,7 @@ public class StorageBuffer {
|
||||
|
||||
}
|
||||
|
||||
|
||||
static class BufferableEntry extends BaseEntry{
|
||||
static class BufferableEntry extends BaseEntry {
|
||||
|
||||
/**
|
||||
*
|
||||
|
@ -2,6 +2,9 @@ package org.apache.lucene.gdata.storage.lucenestorage;
|
||||
|
||||
import java.io.File;
|
||||
import java.io.IOException;
|
||||
import java.util.concurrent.atomic.AtomicBoolean;
|
||||
import java.util.concurrent.locks.Condition;
|
||||
import java.util.concurrent.locks.ReentrantLock;
|
||||
|
||||
import org.apache.commons.logging.Log;
|
||||
import org.apache.commons.logging.LogFactory;
|
||||
@ -42,8 +45,9 @@ public class StorageCoreController implements StorageController {
|
||||
private ReferenceCounter<StorageQuery> storageQuery;
|
||||
|
||||
private StorageBuffer currentBuffer;
|
||||
|
||||
private Object storageControllerLock = new Object();
|
||||
private final AtomicBoolean isClosed = new AtomicBoolean(false);
|
||||
private final ReentrantLock storageControllerLock = new ReentrantLock();
|
||||
private final Condition closeCondition;
|
||||
|
||||
private static final int DEFAULT_STORAGE_BUFFER_SIZE = 10;
|
||||
|
||||
@ -77,6 +81,7 @@ public class StorageCoreController implements StorageController {
|
||||
*/
|
||||
public StorageCoreController() throws IOException, StorageException {
|
||||
synchronized (StorageCoreController.class) {
|
||||
this.closeCondition = this.storageControllerLock.newCondition();
|
||||
try {
|
||||
this.idGenerator = new IDGenerator(10);
|
||||
} catch (Exception e) {
|
||||
@ -168,8 +173,12 @@ public class StorageCoreController implements StorageController {
|
||||
*
|
||||
*/
|
||||
protected ReferenceCounter<StorageQuery> getStorageQuery() {
|
||||
synchronized (this.storageControllerLock) {
|
||||
|
||||
if(this.isClosed.get())
|
||||
throw new IllegalStateException("StorageController is already closed -- server is shutting down");
|
||||
this.storageControllerLock.lock();
|
||||
try{
|
||||
if(this.isClosed.get())
|
||||
throw new IllegalStateException("StorageController is already closed -- server is shutting down");
|
||||
if (this.storageQuery == null) {
|
||||
this.storageQuery = getNewStorageQueryHolder(new StorageQuery(
|
||||
this.currentBuffer, this.searcher));
|
||||
@ -178,6 +187,9 @@ public class StorageCoreController implements StorageController {
|
||||
}
|
||||
this.storageQuery.increamentReference();
|
||||
return this.storageQuery;
|
||||
}finally{
|
||||
this.closeCondition.signalAll();
|
||||
this.storageControllerLock.unlock();
|
||||
}
|
||||
}
|
||||
|
||||
@ -185,6 +197,7 @@ public class StorageCoreController implements StorageController {
|
||||
final StorageQuery query) {
|
||||
ReferenceCounter<StorageQuery> holder = new ReferenceCounter<StorageQuery>(
|
||||
query) {
|
||||
@Override
|
||||
public void close() {
|
||||
try {
|
||||
if (LOG.isInfoEnabled())
|
||||
@ -210,15 +223,24 @@ public class StorageCoreController implements StorageController {
|
||||
* if an IO exception occures
|
||||
*/
|
||||
protected void registerNewStorageQuery() throws IOException {
|
||||
if(this.isClosed.get())
|
||||
throw new IllegalStateException("StorageController is already closed -- server is shutting down");
|
||||
this.storageControllerLock.lock();
|
||||
try{
|
||||
if(this.isClosed.get())
|
||||
throw new IllegalStateException("StorageController is already closed -- server is shutting down");
|
||||
if (LOG.isInfoEnabled())
|
||||
LOG.info("new StorageQuery requested -- create new storage buffer");
|
||||
synchronized (this.storageControllerLock) {
|
||||
|
||||
if (this.storageQuery != null)
|
||||
this.storageQuery.decrementRef();
|
||||
this.searcher = new IndexSearcher(this.storageDir);
|
||||
this.storageQuery = null;
|
||||
this.currentBuffer = new StorageBuffer(this.storageBufferSize);
|
||||
|
||||
}finally{
|
||||
this.closeCondition.signalAll();
|
||||
this.storageControllerLock.unlock();
|
||||
}
|
||||
|
||||
}
|
||||
@ -229,8 +251,16 @@ public class StorageCoreController implements StorageController {
|
||||
* @return the new StorageBuffer
|
||||
*/
|
||||
protected StorageBuffer releaseNewStorageBuffer() {
|
||||
synchronized (this.storageControllerLock) {
|
||||
if(this.isClosed.get())
|
||||
throw new IllegalStateException("StorageController is already closed -- server is shutting down");
|
||||
this.storageControllerLock.lock();
|
||||
try{
|
||||
if(this.isClosed.get())
|
||||
throw new IllegalStateException("StorageController is already closed -- server is shutting down");
|
||||
return this.currentBuffer;
|
||||
}finally{
|
||||
this.closeCondition.signalAll();
|
||||
this.storageControllerLock.unlock();
|
||||
}
|
||||
}
|
||||
|
||||
@ -242,23 +272,45 @@ public class StorageCoreController implements StorageController {
|
||||
* if an IO exception occures
|
||||
*/
|
||||
protected IndexModifier createIndexModifier() throws IOException {
|
||||
if(this.isClosed.get())
|
||||
throw new IllegalStateException("StorageController is already closed -- server is shutting down");
|
||||
this.storageControllerLock.lock();
|
||||
try{
|
||||
if(this.isClosed.get())
|
||||
throw new IllegalStateException("StorageController is already closed -- server is shutting down");
|
||||
if (LOG.isInfoEnabled())
|
||||
LOG.info("new IndexModifier created - release to StorageModifier");
|
||||
synchronized (this.storageControllerLock) {
|
||||
|
||||
return new IndexModifier(this.storageDir, new StandardAnalyzer(),
|
||||
false);
|
||||
}finally{
|
||||
this.closeCondition.signalAll();
|
||||
this.storageControllerLock.unlock();
|
||||
}
|
||||
}
|
||||
|
||||
private void close() throws IOException {
|
||||
synchronized (this.storageControllerLock) {
|
||||
if(this.isClosed.get())
|
||||
throw new IllegalStateException("StorageController is already closed -- server is shutting down");
|
||||
|
||||
this.storageControllerLock.lock();
|
||||
try{
|
||||
if(this.isClosed.get())
|
||||
throw new IllegalStateException("StorageController is already closed -- server is shutting down");
|
||||
this.isClosed.set(true);
|
||||
while(this.storageControllerLock.getQueueLength()>0)
|
||||
try{
|
||||
this.closeCondition.await();
|
||||
}catch (Exception e) {
|
||||
//
|
||||
}
|
||||
if (LOG.isInfoEnabled())
|
||||
LOG
|
||||
.info("StorageController has been closed -- server is shutting down -- release all resources");
|
||||
LOG.info("StorageController has been closed -- server is shutting down -- release all resources");
|
||||
if (this.storageQuery != null)
|
||||
this.storageQuery.decrementRef();
|
||||
this.modifier.close();
|
||||
// TODO make sure all resources will be released
|
||||
}finally{
|
||||
this.storageControllerLock.unlock();
|
||||
}
|
||||
}
|
||||
|
||||
@ -307,6 +359,8 @@ public class StorageCoreController implements StorageController {
|
||||
*
|
||||
*/
|
||||
public void forceWrite() throws IOException {
|
||||
if(this.isClosed.get())
|
||||
throw new IllegalStateException("StorageController is already closed -- server is shutting down");
|
||||
this.modifier.forceWrite();
|
||||
}
|
||||
|
||||
|
@ -7,6 +7,7 @@ import java.util.HashMap;
|
||||
import java.util.LinkedList;
|
||||
import java.util.List;
|
||||
import java.util.Map;
|
||||
import java.util.concurrent.atomic.AtomicBoolean;
|
||||
import java.util.concurrent.locks.Lock;
|
||||
import java.util.concurrent.locks.ReentrantReadWriteLock;
|
||||
|
||||
@ -68,9 +69,11 @@ public class StorageModifier {
|
||||
|
||||
private ReentrantReadWriteLock lock = new ReentrantReadWriteLock(false);
|
||||
|
||||
private Lock readLock = this.lock.readLock();
|
||||
private final AtomicBoolean isClosed = new AtomicBoolean(false);
|
||||
|
||||
private Lock writeLock = this.lock.writeLock();
|
||||
private final Lock readLock = this.lock.readLock();
|
||||
|
||||
private final Lock writeLock = this.lock.writeLock();
|
||||
|
||||
private final static int DEFAULT_OPTIMIZE_INTERVAL = 10;
|
||||
|
||||
@ -315,10 +318,14 @@ public class StorageModifier {
|
||||
}
|
||||
|
||||
private void storageModified() throws StorageException {
|
||||
if(this.isClosed.get())
|
||||
throw new IllegalStateException("StorageModifier is already closed");
|
||||
this.readLock.unlock();
|
||||
this.writeLock.lock();
|
||||
|
||||
try {
|
||||
if(this.isClosed.get())
|
||||
throw new IllegalStateException("StorageModifier is already closed");
|
||||
incrementCounter();
|
||||
if (this.persistFactor > this.modifiedCounter
|
||||
&& this.forceWriteDocuments.size() <= 0
|
||||
@ -345,6 +352,8 @@ public class StorageModifier {
|
||||
}
|
||||
|
||||
protected void forceWrite() throws IOException {
|
||||
if(this.isClosed.get())
|
||||
throw new IllegalStateException("StorageModifier is already closed");
|
||||
this.writeLock.lock();
|
||||
try {
|
||||
if (LOG.isInfoEnabled())
|
||||
@ -424,8 +433,13 @@ public class StorageModifier {
|
||||
}
|
||||
|
||||
protected void close() throws IOException {
|
||||
if(this.isClosed.get())
|
||||
throw new IllegalStateException("StorageModifier is already closed");
|
||||
this.writeLock.lock();
|
||||
try {
|
||||
if(this.isClosed.get())
|
||||
throw new IllegalStateException("StorageModifier is already closed");
|
||||
this.isClosed.set(true);
|
||||
if (LOG.isInfoEnabled())
|
||||
LOG.info("ForceWrite called -- current modifiedCounter: "
|
||||
+ this.modifiedCounter + " - persisting changes");
|
||||
|
@ -19,6 +19,7 @@ package org.apache.lucene.gdata.utils;
|
||||
import java.text.ParseException;
|
||||
import java.text.SimpleDateFormat;
|
||||
import java.util.Date;
|
||||
import java.util.Locale;
|
||||
import java.util.Stack;
|
||||
|
||||
/**
|
||||
@ -106,6 +107,7 @@ public class DateFormater {
|
||||
if(dateString == null|| pattern == null)
|
||||
throw new IllegalArgumentException(
|
||||
"given parameters must not be null");
|
||||
|
||||
SimpleDateFormat inst = formater.getFormater();
|
||||
inst.applyPattern(pattern);
|
||||
return inst.parse(dateString);
|
||||
@ -113,7 +115,7 @@ public class DateFormater {
|
||||
|
||||
private SimpleDateFormat getFormater() {
|
||||
if (this.objectStack.empty())
|
||||
return new SimpleDateFormat();
|
||||
return new SimpleDateFormat(DateFormater.HTTP_HEADER_DATE_FORMAT,Locale.ENGLISH);
|
||||
return this.objectStack.pop();
|
||||
}
|
||||
|
||||
|
@ -55,11 +55,11 @@ public class TestModifiedEntryFilter extends TestCase {
|
||||
Hits hits = s.search(q);
|
||||
assertEquals(2,hits.length());
|
||||
|
||||
hits = s.search(q,new ModifiedEntryFilter(this.excludeList));
|
||||
hits = s.search(q,new ModifiedEntryFilter(this.excludeList.toArray(new String[0])));
|
||||
assertEquals(1,hits.length());
|
||||
this.excludeList.add("2");
|
||||
|
||||
hits = s.search(q,new ModifiedEntryFilter(this.excludeList));
|
||||
hits = s.search(q,new ModifiedEntryFilter(this.excludeList.toArray(new String[0])));
|
||||
assertEquals(0,hits.length());
|
||||
|
||||
}
|
||||
|
@ -136,16 +136,27 @@ public class TestStorageModifier extends TestCase {
|
||||
|
||||
Thread b = getRunnerThread((this.count += 10));
|
||||
b.start();
|
||||
// wait for the first thread to check for the inserted entries
|
||||
a.join();
|
||||
try{
|
||||
for (int i = 1; i < this.count; i++) {
|
||||
ReferenceCounter<StorageQuery> innerQuery = this.controller
|
||||
.getStorageQuery();
|
||||
BaseEntry e = innerQuery.get().singleEntryQuery("" + i, feedId,
|
||||
this.configurator);
|
||||
assertNotNull(e);
|
||||
assertEquals("get entry for id" + i, "" + i, e.getId());
|
||||
|
||||
}
|
||||
}finally{
|
||||
/*
|
||||
* if an exception occures the tread can at least finnish running before the
|
||||
* controller will be closed in the tearDown method
|
||||
*/
|
||||
b.join();
|
||||
}
|
||||
|
||||
|
||||
ReferenceCounter<StorageQuery> query = this.controller
|
||||
.getStorageQuery();
|
||||
|
||||
|
@ -25,13 +25,16 @@ public class TestDateFormater extends TestCase {
|
||||
public void testFormatDate() throws ParseException {
|
||||
|
||||
// this reg. --> bit weak but does the job
|
||||
java.util.regex.Pattern pattern = java.util.regex.Pattern.compile("[A-Z][a-z]{2}, [0-9]{1,2} [A-Z][a-z]{2} [0-9]{4} [0-9]{1,2}:[0-9]{1,2}:[0-9]{1,2} [A-Z]{2,4}");
|
||||
java.util.regex.Pattern pattern = java.util.regex.Pattern.compile("[A-Z][a-z]{1,2}, [0-9]{1,2} [A-Z][a-z]{2} [0-9]{4} [0-9]{1,2}:[0-9]{1,2}:[0-9]{1,2} [A-Z]{2,4}");
|
||||
Date date = new Date();
|
||||
System.out.println(date.getTime());
|
||||
|
||||
String formatedDate = DateFormater.formatDate(date,DateFormater.HTTP_HEADER_DATE_FORMAT);
|
||||
|
||||
assertTrue(pattern.matcher(formatedDate).matches());
|
||||
System.out.println(DateFormater.parseDate(formatedDate,DateFormater.HTTP_HEADER_DATE_FORMAT).getTime());
|
||||
DateFormater.parseDate("Sun, 25 Jun 2006 13:51:23 +0000",DateFormater.HTTP_HEADER_DATE_FORMAT,DateFormater.HTTP_HEADER_DATE_FORMAT_TIME_OFFSET);
|
||||
|
||||
DateFormater.parseDate("Sun, 25 Jun 2006 13:51:23 CEST",DateFormater.HTTP_HEADER_DATE_FORMAT,DateFormater.HTTP_HEADER_DATE_FORMAT_TIME_OFFSET);
|
||||
//TODO extend this
|
||||
|
||||
|
||||
}
|
||||
|
||||
|
Loading…
x
Reference in New Issue
Block a user