mirror of https://github.com/apache/lucene.git
Merge remote-tracking branch 'upstream/master' into SOLR-10335
This commit is contained in:
commit
4c7ff73c98
|
@ -161,7 +161,7 @@
|
|||
<plugin>
|
||||
<groupId>de.thetaphi</groupId>
|
||||
<artifactId>forbiddenapis</artifactId>
|
||||
<version>2.2</version>
|
||||
<version>2.4.1</version>
|
||||
<configuration>
|
||||
<!--
|
||||
This is the default setting, we don't support too new Java versions.
|
||||
|
|
|
@ -210,7 +210,7 @@ def checkAll(dirName):
|
|||
elif link.find('lucene.apache.org/solr/mirrors-solr-latest-redir.html') != -1:
|
||||
# OK
|
||||
pass
|
||||
elif link.find('lucene.apache.org/solr/quickstart.html') != -1:
|
||||
elif link.find('lucene.apache.org/solr/guide/') != -1:
|
||||
# OK
|
||||
pass
|
||||
elif link.find('lucene.apache.org/solr/downloads.html') != -1:
|
||||
|
|
|
@ -39,7 +39,14 @@ New Features
|
|||
K-nearest-neighbor search implementation. (Steve Rowe)
|
||||
|
||||
* LUCENE-7975: Change the default taxonomy facets cache to a faster
|
||||
byte[] (UTF-8) based cache.
|
||||
byte[] (UTF-8) based cache. (Mike McCandless)
|
||||
|
||||
* LUCENE-7972: DirectoryTaxonomyReader, in Lucene's facet module, now
|
||||
implements Accountable, so you can more easily track how much heap
|
||||
it's using. (Mike McCandless)
|
||||
|
||||
* LUCENE-7982: A new NormsFieldExistsQuery matches documents that have
|
||||
norms in a specified field (Colin Goodheart-Smithe via Mike McCandless)
|
||||
|
||||
Optimizations
|
||||
|
||||
|
|
|
@ -0,0 +1,80 @@
|
|||
/*
|
||||
* Licensed to the Apache Software Foundation (ASF) under one or more
|
||||
* contributor license agreements. See the NOTICE file distributed with
|
||||
* this work for additional information regarding copyright ownership.
|
||||
* The ASF 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 org.apache.lucene.search;
|
||||
|
||||
|
||||
import java.io.IOException;
|
||||
import java.util.Objects;
|
||||
|
||||
import org.apache.lucene.document.StringField;
|
||||
import org.apache.lucene.index.FieldInfo;
|
||||
import org.apache.lucene.index.FieldInfos;
|
||||
import org.apache.lucene.index.LeafReader;
|
||||
import org.apache.lucene.index.LeafReaderContext;
|
||||
|
||||
/**
|
||||
* A {@link Query} that matches documents that have a value for a given field
|
||||
* as reported by field norms. This will not work for fields that omit norms,
|
||||
* e.g. {@link StringField}.
|
||||
*/
|
||||
public final class NormsFieldExistsQuery extends Query {
|
||||
|
||||
private final String field;
|
||||
|
||||
/** Create a query that will match that have a value for the given
|
||||
* {@code field}. */
|
||||
public NormsFieldExistsQuery(String field) {
|
||||
this.field = Objects.requireNonNull(field);
|
||||
}
|
||||
|
||||
public String getField() {
|
||||
return field;
|
||||
}
|
||||
|
||||
@Override
|
||||
public boolean equals(Object other) {
|
||||
return sameClassAs(other) &&
|
||||
field.equals(((NormsFieldExistsQuery) other).field);
|
||||
}
|
||||
|
||||
@Override
|
||||
public int hashCode() {
|
||||
return 31 * classHash() + field.hashCode();
|
||||
}
|
||||
|
||||
@Override
|
||||
public String toString(String field) {
|
||||
return "NormsFieldExistsQuery [field=" + this.field + "]";
|
||||
}
|
||||
|
||||
@Override
|
||||
public Weight createWeight(IndexSearcher searcher, boolean needsScores, float boost) throws IOException {
|
||||
return new ConstantScoreWeight(this, boost) {
|
||||
@Override
|
||||
public Scorer scorer(LeafReaderContext context) throws IOException {
|
||||
FieldInfos fieldInfos = context.reader().getFieldInfos();
|
||||
FieldInfo fieldInfo = fieldInfos.fieldInfo(field);
|
||||
if (fieldInfo == null || fieldInfo.hasNorms() == false) {
|
||||
return null;
|
||||
}
|
||||
LeafReader reader = context.reader();
|
||||
DocIdSetIterator iterator = reader.getNormValues(field);
|
||||
return new ConstantScoreScorer(this, score(), iterator);
|
||||
}
|
||||
};
|
||||
}
|
||||
}
|
|
@ -0,0 +1,197 @@
|
|||
/*
|
||||
* Licensed to the Apache Software Foundation (ASF) under one or more
|
||||
* contributor license agreements. See the NOTICE file distributed with
|
||||
* this work for additional information regarding copyright ownership.
|
||||
* The ASF 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 org.apache.lucene.search;
|
||||
|
||||
|
||||
import java.io.IOException;
|
||||
|
||||
import org.apache.lucene.document.Document;
|
||||
import org.apache.lucene.document.Field.Store;
|
||||
import org.apache.lucene.document.StringField;
|
||||
import org.apache.lucene.document.TextField;
|
||||
import org.apache.lucene.index.IndexReader;
|
||||
import org.apache.lucene.index.RandomIndexWriter;
|
||||
import org.apache.lucene.index.Term;
|
||||
import org.apache.lucene.search.BooleanClause.Occur;
|
||||
import org.apache.lucene.store.Directory;
|
||||
import org.apache.lucene.util.LuceneTestCase;
|
||||
|
||||
public class TestNormsFieldExistsQuery extends LuceneTestCase {
|
||||
|
||||
public void testRandom() throws IOException {
|
||||
final int iters = atLeast(10);
|
||||
for (int iter = 0; iter < iters; ++iter) {
|
||||
Directory dir = newDirectory();
|
||||
RandomIndexWriter iw = new RandomIndexWriter(random(), dir);
|
||||
final int numDocs = atLeast(100);
|
||||
for (int i = 0; i < numDocs; ++i) {
|
||||
Document doc = new Document();
|
||||
final boolean hasValue = random().nextBoolean();
|
||||
if (hasValue) {
|
||||
doc.add(new TextField("text1", "value", Store.NO));
|
||||
doc.add(new StringField("has_value", "yes", Store.NO));
|
||||
}
|
||||
doc.add(new StringField("f", random().nextBoolean() ? "yes" : "no", Store.NO));
|
||||
iw.addDocument(doc);
|
||||
}
|
||||
if (random().nextBoolean()) {
|
||||
iw.deleteDocuments(new TermQuery(new Term("f", "no")));
|
||||
}
|
||||
iw.commit();
|
||||
final IndexReader reader = iw.getReader();
|
||||
final IndexSearcher searcher = newSearcher(reader);
|
||||
iw.close();
|
||||
|
||||
assertSameMatches(searcher, new TermQuery(new Term("has_value", "yes")), new NormsFieldExistsQuery("text1"), false);
|
||||
|
||||
reader.close();
|
||||
dir.close();
|
||||
}
|
||||
}
|
||||
|
||||
public void testApproximation() throws IOException {
|
||||
final int iters = atLeast(10);
|
||||
for (int iter = 0; iter < iters; ++iter) {
|
||||
Directory dir = newDirectory();
|
||||
RandomIndexWriter iw = new RandomIndexWriter(random(), dir);
|
||||
final int numDocs = atLeast(100);
|
||||
for (int i = 0; i < numDocs; ++i) {
|
||||
Document doc = new Document();
|
||||
final boolean hasValue = random().nextBoolean();
|
||||
if (hasValue) {
|
||||
doc.add(new TextField("text1", "value", Store.NO));
|
||||
doc.add(new StringField("has_value", "yes", Store.NO));
|
||||
}
|
||||
doc.add(new StringField("f", random().nextBoolean() ? "yes" : "no", Store.NO));
|
||||
iw.addDocument(doc);
|
||||
}
|
||||
if (random().nextBoolean()) {
|
||||
iw.deleteDocuments(new TermQuery(new Term("f", "no")));
|
||||
}
|
||||
iw.commit();
|
||||
final IndexReader reader = iw.getReader();
|
||||
final IndexSearcher searcher = newSearcher(reader);
|
||||
iw.close();
|
||||
|
||||
BooleanQuery.Builder ref = new BooleanQuery.Builder();
|
||||
ref.add(new TermQuery(new Term("f", "yes")), Occur.MUST);
|
||||
ref.add(new TermQuery(new Term("has_value", "yes")), Occur.FILTER);
|
||||
|
||||
BooleanQuery.Builder bq1 = new BooleanQuery.Builder();
|
||||
bq1.add(new TermQuery(new Term("f", "yes")), Occur.MUST);
|
||||
bq1.add(new NormsFieldExistsQuery("text1"), Occur.FILTER);
|
||||
assertSameMatches(searcher, ref.build(), bq1.build(), true);
|
||||
|
||||
reader.close();
|
||||
dir.close();
|
||||
}
|
||||
}
|
||||
|
||||
public void testScore() throws IOException {
|
||||
final int iters = atLeast(10);
|
||||
for (int iter = 0; iter < iters; ++iter) {
|
||||
Directory dir = newDirectory();
|
||||
RandomIndexWriter iw = new RandomIndexWriter(random(), dir);
|
||||
final int numDocs = atLeast(100);
|
||||
for (int i = 0; i < numDocs; ++i) {
|
||||
Document doc = new Document();
|
||||
final boolean hasValue = random().nextBoolean();
|
||||
if (hasValue) {
|
||||
doc.add(new TextField("text1", "value", Store.NO));
|
||||
doc.add(new StringField("has_value", "yes", Store.NO));
|
||||
}
|
||||
doc.add(new StringField("f", random().nextBoolean() ? "yes" : "no", Store.NO));
|
||||
iw.addDocument(doc);
|
||||
}
|
||||
if (random().nextBoolean()) {
|
||||
iw.deleteDocuments(new TermQuery(new Term("f", "no")));
|
||||
}
|
||||
iw.commit();
|
||||
final IndexReader reader = iw.getReader();
|
||||
final IndexSearcher searcher = newSearcher(reader);
|
||||
iw.close();
|
||||
|
||||
final float boost = random().nextFloat() * 10;
|
||||
final Query ref = new BoostQuery(new ConstantScoreQuery(new TermQuery(new Term("has_value", "yes"))), boost);
|
||||
|
||||
final Query q1 = new BoostQuery(new NormsFieldExistsQuery("text1"), boost);
|
||||
assertSameMatches(searcher, ref, q1, true);
|
||||
|
||||
reader.close();
|
||||
dir.close();
|
||||
}
|
||||
}
|
||||
|
||||
public void testMissingField() throws IOException {
|
||||
Directory dir = newDirectory();
|
||||
RandomIndexWriter iw = new RandomIndexWriter(random(), dir);
|
||||
iw.addDocument(new Document());
|
||||
iw.commit();
|
||||
final IndexReader reader = iw.getReader();
|
||||
final IndexSearcher searcher = newSearcher(reader);
|
||||
iw.close();
|
||||
assertEquals(0, searcher.search(new NormsFieldExistsQuery("f"), 1).totalHits);
|
||||
reader.close();
|
||||
dir.close();
|
||||
}
|
||||
|
||||
public void testAllDocsHaveField() throws IOException {
|
||||
Directory dir = newDirectory();
|
||||
RandomIndexWriter iw = new RandomIndexWriter(random(), dir);
|
||||
Document doc = new Document();
|
||||
doc.add(new TextField("f", "value", Store.NO));
|
||||
iw.addDocument(doc);
|
||||
iw.commit();
|
||||
final IndexReader reader = iw.getReader();
|
||||
final IndexSearcher searcher = newSearcher(reader);
|
||||
iw.close();
|
||||
assertEquals(1, searcher.search(new NormsFieldExistsQuery("f"), 1).totalHits);
|
||||
reader.close();
|
||||
dir.close();
|
||||
}
|
||||
|
||||
public void testFieldExistsButNoDocsHaveField() throws IOException {
|
||||
Directory dir = newDirectory();
|
||||
RandomIndexWriter iw = new RandomIndexWriter(random(), dir);
|
||||
// 1st segment has the field, but 2nd one does not
|
||||
Document doc = new Document();
|
||||
doc.add(new TextField("f", "value", Store.NO));
|
||||
iw.addDocument(doc);
|
||||
iw.commit();
|
||||
iw.addDocument(new Document());
|
||||
iw.commit();
|
||||
final IndexReader reader = iw.getReader();
|
||||
final IndexSearcher searcher = newSearcher(reader);
|
||||
iw.close();
|
||||
assertEquals(1, searcher.search(new NormsFieldExistsQuery("f"), 1).totalHits);
|
||||
reader.close();
|
||||
dir.close();
|
||||
}
|
||||
|
||||
private void assertSameMatches(IndexSearcher searcher, Query q1, Query q2, boolean scores) throws IOException {
|
||||
final int maxDoc = searcher.getIndexReader().maxDoc();
|
||||
final TopDocs td1 = searcher.search(q1, maxDoc, scores ? Sort.RELEVANCE : Sort.INDEXORDER);
|
||||
final TopDocs td2 = searcher.search(q2, maxDoc, scores ? Sort.RELEVANCE : Sort.INDEXORDER);
|
||||
assertEquals(td1.totalHits, td2.totalHits);
|
||||
for (int i = 0; i < td1.scoreDocs.length; ++i) {
|
||||
assertEquals(td1.scoreDocs[i].doc, td2.scoreDocs[i].doc);
|
||||
if (scores) {
|
||||
assertEquals(td1.scoreDocs[i].score, td2.scoreDocs[i].score, 10e-7);
|
||||
}
|
||||
}
|
||||
}
|
||||
}
|
|
@ -17,6 +17,10 @@
|
|||
package org.apache.lucene.facet.taxonomy.directory;
|
||||
|
||||
import java.io.IOException;
|
||||
import java.util.ArrayList;
|
||||
import java.util.Collection;
|
||||
import java.util.Collections;
|
||||
import java.util.List;
|
||||
import java.util.Map;
|
||||
import java.util.logging.Level;
|
||||
import java.util.logging.Logger;
|
||||
|
@ -30,12 +34,17 @@ import org.apache.lucene.facet.taxonomy.TaxonomyReader;
|
|||
import org.apache.lucene.index.CorruptIndexException; // javadocs
|
||||
import org.apache.lucene.index.DirectoryReader;
|
||||
import org.apache.lucene.index.IndexWriter;
|
||||
import org.apache.lucene.index.LeafReaderContext;
|
||||
import org.apache.lucene.index.MultiFields;
|
||||
import org.apache.lucene.index.PostingsEnum;
|
||||
import org.apache.lucene.index.SegmentReader;
|
||||
import org.apache.lucene.search.DocIdSetIterator;
|
||||
import org.apache.lucene.store.Directory;
|
||||
import org.apache.lucene.util.Accountable;
|
||||
import org.apache.lucene.util.Accountables;
|
||||
import org.apache.lucene.util.BytesRef;
|
||||
import org.apache.lucene.util.IOUtils;
|
||||
import org.apache.lucene.util.RamUsageEstimator;
|
||||
|
||||
/**
|
||||
* A {@link TaxonomyReader} which retrieves stored taxonomy information from a
|
||||
|
@ -49,11 +58,14 @@ import org.apache.lucene.util.IOUtils;
|
|||
*
|
||||
* @lucene.experimental
|
||||
*/
|
||||
public class DirectoryTaxonomyReader extends TaxonomyReader {
|
||||
public class DirectoryTaxonomyReader extends TaxonomyReader implements Accountable {
|
||||
|
||||
private static final Logger logger = Logger.getLogger(DirectoryTaxonomyReader.class.getName());
|
||||
|
||||
private static final int DEFAULT_CACHE_VALUE = 4000;
|
||||
|
||||
// NOTE: very coarse estimate!
|
||||
private static final int BYTES_PER_CACHE_ENTRY = 4 * RamUsageEstimator.NUM_BYTES_OBJECT_REF + 4 * RamUsageEstimator.NUM_BYTES_OBJECT_HEADER + 8 * RamUsageEstimator.NUM_BYTES_CHAR;
|
||||
|
||||
private final DirectoryTaxonomyWriter taxoWriter;
|
||||
private final long taxoEpoch; // used in doOpenIfChanged
|
||||
|
@ -325,7 +337,51 @@ public class DirectoryTaxonomyReader extends TaxonomyReader {
|
|||
ensureOpen();
|
||||
return indexReader.numDocs();
|
||||
}
|
||||
|
||||
@Override
|
||||
public synchronized long ramBytesUsed() {
|
||||
ensureOpen();
|
||||
long ramBytesUsed = 0;
|
||||
for (LeafReaderContext ctx : indexReader.leaves()) {
|
||||
ramBytesUsed += ((SegmentReader) ctx.reader()).ramBytesUsed();
|
||||
}
|
||||
if (taxoArrays != null) {
|
||||
ramBytesUsed += taxoArrays.ramBytesUsed();
|
||||
}
|
||||
synchronized (categoryCache) {
|
||||
ramBytesUsed += BYTES_PER_CACHE_ENTRY * categoryCache.size();
|
||||
}
|
||||
|
||||
synchronized (ordinalCache) {
|
||||
ramBytesUsed += BYTES_PER_CACHE_ENTRY * ordinalCache.size();
|
||||
}
|
||||
|
||||
return ramBytesUsed;
|
||||
}
|
||||
|
||||
@Override
|
||||
public synchronized Collection<Accountable> getChildResources() {
|
||||
final List<Accountable> resources = new ArrayList<>();
|
||||
long ramBytesUsed = 0;
|
||||
for (LeafReaderContext ctx : indexReader.leaves()) {
|
||||
ramBytesUsed += ((SegmentReader) ctx.reader()).ramBytesUsed();
|
||||
}
|
||||
resources.add(Accountables.namedAccountable("indexReader", ramBytesUsed));
|
||||
if (taxoArrays != null) {
|
||||
resources.add(Accountables.namedAccountable("taxoArrays", taxoArrays));
|
||||
}
|
||||
|
||||
synchronized (categoryCache) {
|
||||
resources.add(Accountables.namedAccountable("categoryCache", BYTES_PER_CACHE_ENTRY * categoryCache.size()));
|
||||
}
|
||||
|
||||
synchronized (ordinalCache) {
|
||||
resources.add(Accountables.namedAccountable("ordinalCache", BYTES_PER_CACHE_ENTRY * ordinalCache.size()));
|
||||
}
|
||||
|
||||
return Collections.unmodifiableList(resources);
|
||||
}
|
||||
|
||||
/**
|
||||
* setCacheSize controls the maximum allowed size of each of the caches
|
||||
* used by {@link #getPath(int)} and {@link #getOrdinal(FacetLabel)}.
|
||||
|
|
|
@ -16,16 +16,23 @@
|
|||
*/
|
||||
package org.apache.lucene.facet.taxonomy.directory;
|
||||
|
||||
import java.io.IOException;
|
||||
import java.util.ArrayList;
|
||||
import java.util.Collection;
|
||||
import java.util.Collections;
|
||||
import java.util.List;
|
||||
|
||||
import org.apache.lucene.facet.taxonomy.ParallelTaxonomyArrays;
|
||||
import org.apache.lucene.facet.taxonomy.TaxonomyReader;
|
||||
import org.apache.lucene.index.CorruptIndexException;
|
||||
import org.apache.lucene.index.PostingsEnum;
|
||||
import org.apache.lucene.index.IndexReader;
|
||||
import org.apache.lucene.index.MultiFields;
|
||||
import org.apache.lucene.index.PostingsEnum;
|
||||
import org.apache.lucene.search.DocIdSetIterator;
|
||||
import org.apache.lucene.util.Accountable;
|
||||
import org.apache.lucene.util.Accountables;
|
||||
import org.apache.lucene.util.ArrayUtil;
|
||||
|
||||
import java.io.IOException;
|
||||
import org.apache.lucene.util.RamUsageEstimator;
|
||||
|
||||
/**
|
||||
* A {@link ParallelTaxonomyArrays} that are initialized from the taxonomy
|
||||
|
@ -33,7 +40,7 @@ import java.io.IOException;
|
|||
*
|
||||
* @lucene.experimental
|
||||
*/
|
||||
class TaxonomyIndexArrays extends ParallelTaxonomyArrays {
|
||||
class TaxonomyIndexArrays extends ParallelTaxonomyArrays implements Accountable {
|
||||
|
||||
private final int[] parents;
|
||||
|
||||
|
@ -214,4 +221,29 @@ class TaxonomyIndexArrays extends ParallelTaxonomyArrays {
|
|||
return siblings;
|
||||
}
|
||||
|
||||
@Override
|
||||
public synchronized long ramBytesUsed() {
|
||||
long ramBytesUsed = RamUsageEstimator.NUM_BYTES_OBJECT_HEADER + 3 * RamUsageEstimator.NUM_BYTES_OBJECT_REF + RamUsageEstimator.NUM_BYTES_BOOLEAN;
|
||||
ramBytesUsed += RamUsageEstimator.shallowSizeOf(parents);
|
||||
if (children != null) {
|
||||
ramBytesUsed += RamUsageEstimator.shallowSizeOf(children);
|
||||
}
|
||||
if (siblings != null) {
|
||||
ramBytesUsed += RamUsageEstimator.shallowSizeOf(siblings);
|
||||
}
|
||||
return ramBytesUsed;
|
||||
}
|
||||
|
||||
@Override
|
||||
public synchronized Collection<Accountable> getChildResources() {
|
||||
final List<Accountable> resources = new ArrayList<>();
|
||||
resources.add(Accountables.namedAccountable("parents", RamUsageEstimator.shallowSizeOf(parents)));
|
||||
if (children != null) {
|
||||
resources.add(Accountables.namedAccountable("children", RamUsageEstimator.shallowSizeOf(children)));
|
||||
}
|
||||
if (siblings != null) {
|
||||
resources.add(Accountables.namedAccountable("siblings", RamUsageEstimator.shallowSizeOf(siblings)));
|
||||
}
|
||||
return Collections.unmodifiableList(resources);
|
||||
}
|
||||
}
|
||||
|
|
|
@ -25,12 +25,12 @@ import java.util.Set;
|
|||
import org.apache.lucene.analysis.MockAnalyzer;
|
||||
import org.apache.lucene.facet.FacetTestCase;
|
||||
import org.apache.lucene.facet.taxonomy.FacetLabel;
|
||||
import org.apache.lucene.facet.taxonomy.TaxonomyReader;
|
||||
import org.apache.lucene.facet.taxonomy.TaxonomyReader.ChildrenIterator;
|
||||
import org.apache.lucene.facet.taxonomy.TaxonomyReader;
|
||||
import org.apache.lucene.facet.taxonomy.TaxonomyWriter;
|
||||
import org.apache.lucene.index.IndexWriter;
|
||||
import org.apache.lucene.index.IndexWriterConfig;
|
||||
import org.apache.lucene.index.IndexWriterConfig.OpenMode;
|
||||
import org.apache.lucene.index.IndexWriterConfig;
|
||||
import org.apache.lucene.index.LogByteSizeMergePolicy;
|
||||
import org.apache.lucene.index.LogMergePolicy;
|
||||
import org.apache.lucene.store.AlreadyClosedException;
|
||||
|
@ -529,5 +529,33 @@ public class TestDirectoryTaxonomyReader extends FacetTestCase {
|
|||
|
||||
dir.close();
|
||||
}
|
||||
|
||||
|
||||
public void testAccountable() throws Exception {
|
||||
Directory dir = newDirectory();
|
||||
DirectoryTaxonomyWriter taxoWriter = new DirectoryTaxonomyWriter(dir);
|
||||
int numCategories = atLeast(10);
|
||||
int numA = 0, numB = 0;
|
||||
Random random = random();
|
||||
// add the two categories for which we'll also add children (so asserts are simpler)
|
||||
taxoWriter.addCategory(new FacetLabel("a"));
|
||||
taxoWriter.addCategory(new FacetLabel("b"));
|
||||
for (int i = 0; i < numCategories; i++) {
|
||||
if (random.nextBoolean()) {
|
||||
taxoWriter.addCategory(new FacetLabel("a", Integer.toString(i)));
|
||||
++numA;
|
||||
} else {
|
||||
taxoWriter.addCategory(new FacetLabel("b", Integer.toString(i)));
|
||||
++numB;
|
||||
}
|
||||
}
|
||||
// add category with no children
|
||||
taxoWriter.addCategory(new FacetLabel("c"));
|
||||
taxoWriter.close();
|
||||
|
||||
DirectoryTaxonomyReader taxoReader = new DirectoryTaxonomyReader(dir);
|
||||
assertTrue(taxoReader.ramBytesUsed() > 0);
|
||||
assertTrue(taxoReader.getChildResources().size() > 0);
|
||||
taxoReader.close();
|
||||
dir.close();
|
||||
}
|
||||
}
|
||||
|
|
|
@ -14,7 +14,7 @@ Getting Started
|
|||
You need a Java 1.8 VM or later installed.
|
||||
In this release, there is an example Solr server including a bundled
|
||||
servlet container in the directory named "example".
|
||||
See the Quick Start guide at http://lucene.apache.org/solr/quickstart.html
|
||||
See the Solr tutorial at https://lucene.apache.org/solr/guide/solr-tutorial.html
|
||||
|
||||
================== 8.0.0 ==================
|
||||
|
||||
|
|
|
@ -87,8 +87,8 @@ For more information about Solr examples please read...
|
|||
|
||||
* example/README.txt
|
||||
For more information about the "Solr Home" and Solr specific configuration
|
||||
* http://lucene.apache.org/solr/quickstart.html
|
||||
For a Quick Start guide
|
||||
* https://lucene.apache.org/solr/guide/solr-tutorial.html
|
||||
For a Solr tutorial
|
||||
* http://lucene.apache.org/solr/resources.html
|
||||
For a list of other tutorials and introductory articles.
|
||||
|
||||
|
|
|
@ -191,22 +191,6 @@
|
|||
depends="javadocs,changes-to-html,process-webpages"/>
|
||||
<target name="compile-core" depends="compile-solr-core" unless="solr.core.compiled"/>
|
||||
|
||||
<target name="generate-website-quickstart"
|
||||
description="Generate a version of the quickstart tutorial suitable for the website, at build/website/quickstart.mdtext">
|
||||
<copy file="${common-solr.dir}/site/quickstart.mdtext" tofile="${common-solr.dir}/build/website/quickstart.mdtext"
|
||||
overwrite="false" encoding="UTF-8">
|
||||
<filterchain>
|
||||
<tokenfilter>
|
||||
<filetokenizer/>
|
||||
<!-- Website images are under /solr/assets/images/ -->
|
||||
<replaceregex pattern="src\s*=\s*"images/" replace="src="/solr/assets/images/" flags="gs"/>
|
||||
<!-- Redirect to the website's version-specific system requirements page -->
|
||||
<replaceregex pattern="\(SYSTEM_REQUIREMENTS.html\)" replace="(/solr/api/SYSTEM_REQUIREMENTS.html)" flags="gs"/>
|
||||
</tokenfilter>
|
||||
</filterchain>
|
||||
</copy>
|
||||
</target>
|
||||
|
||||
<target name="documentation-online" description="Generate a link to the online documentation"
|
||||
depends="define-solr-javadoc-url">
|
||||
<xslt in="${ant.file}" out="${javadoc-online.dir}/index.html" style="site/online-link.xsl" force="true">
|
||||
|
@ -226,6 +210,16 @@
|
|||
<makeurl property="process-webpages.buildfiles" separator="|">
|
||||
<fileset dir="." includes="core/build.xml,test-framework/build.xml,solrj/build.xml,contrib/**/build.xml"/>
|
||||
</makeurl>
|
||||
|
||||
<loadresource property="doc-solr-guide-version-path">
|
||||
<propertyresource name="version"/>
|
||||
<filterchain>
|
||||
<tokenfilter>
|
||||
<filetokenizer/>
|
||||
<replaceregex pattern="^(\d+)\.(\d+).*" replace="\1_\2"/>
|
||||
</tokenfilter>
|
||||
</filterchain>
|
||||
</loadresource>
|
||||
<!--
|
||||
The XSL input file is ignored completely, but XSL expects one to be given,
|
||||
so we pass ourself (${ant.file}) here. The list of module build.xmls is given
|
||||
|
@ -239,6 +233,7 @@
|
|||
<param name="buildfiles" expression="${process-webpages.buildfiles}"/>
|
||||
<param name="version" expression="${version}"/>
|
||||
<param name="luceneJavadocUrl" expression="${lucene.javadoc.url}"/>
|
||||
<param name="solrGuideVersion" expression="${doc-solr-guide-version-path}"/>
|
||||
</xslt>
|
||||
|
||||
<markdown todir="${javadoc.dir}">
|
||||
|
@ -676,7 +671,7 @@
|
|||
<!-- NOTE: must currently exclude deprecated-list due to a javadocs bug (as of 1.7.0_09)
|
||||
javadocs generates invalid XML if you deprecate a method that takes a parameter
|
||||
with a generic type -->
|
||||
<fileset dir="build/docs" includes="**/*.html" excludes="**/deprecated-list.html,quickstart.html"/>
|
||||
<fileset dir="build/docs" includes="**/*.html" excludes="**/deprecated-list.html"/>
|
||||
</jtidy-macro>
|
||||
<echo message="Checking for broken links..."/>
|
||||
<check-broken-links dir="${javadoc.dir}"/>
|
||||
|
|
|
@ -14,7 +14,7 @@ For information on how to get started with solr ltr please see:
|
|||
|
||||
For information on how to get started with solr please see:
|
||||
* [solr/README.txt](../../README.txt)
|
||||
* [Solr Quick Start](http://lucene.apache.org/solr/quickstart.html)
|
||||
* [Solr Tutorial](https://lucene.apache.org/solr/guide/solr-tutorial.html)
|
||||
|
||||
# How To Contribute
|
||||
|
||||
|
|
|
@ -24,7 +24,6 @@ import java.util.Collections;
|
|||
import java.util.HashMap;
|
||||
import java.util.Locale;
|
||||
import java.util.Map;
|
||||
import java.util.concurrent.atomic.AtomicBoolean;
|
||||
import java.util.concurrent.atomic.AtomicLong;
|
||||
|
||||
import org.apache.commons.lang.StringUtils;
|
||||
|
|
|
@ -23,7 +23,6 @@ import java.util.List;
|
|||
import java.util.Map;
|
||||
import java.util.Objects;
|
||||
|
||||
import com.google.common.base.Preconditions;
|
||||
import org.apache.lucene.store.AlreadyClosedException;
|
||||
import org.apache.solr.client.solrj.cloud.autoscaling.TriggerEventType;
|
||||
import org.apache.solr.common.util.Utils;
|
||||
|
@ -50,7 +49,7 @@ public class AutoScaling {
|
|||
/**
|
||||
* Interface for a Solr trigger. Each trigger implements Runnable and Closeable interface. A trigger
|
||||
* is scheduled using a {@link java.util.concurrent.ScheduledExecutorService} so it is executed as
|
||||
* per a configured schedule to check whether the trigger is ready to fire. The {@link Trigger#setProcessor(TriggerEventProcessor)}
|
||||
* per a configured schedule to check whether the trigger is ready to fire. The {@link AutoScaling.Trigger#setProcessor(AutoScaling.TriggerEventProcessor)}
|
||||
* method should be used to set a processor which is used by implementation of this class whenever
|
||||
* ready.
|
||||
* <p>
|
||||
|
|
|
@ -146,7 +146,22 @@ public class ScheduledTriggers implements Closeable {
|
|||
if (isClosed) {
|
||||
throw new AlreadyClosedException("ScheduledTriggers has been closed and cannot be used anymore");
|
||||
}
|
||||
ScheduledTrigger scheduledTrigger = new ScheduledTrigger(newTrigger, zkClient, queueStats);
|
||||
ScheduledTrigger st;
|
||||
try {
|
||||
st = new ScheduledTrigger(newTrigger, zkClient, queueStats);
|
||||
} catch (Exception e) {
|
||||
if (isClosed) {
|
||||
throw new AlreadyClosedException("ScheduledTriggers has been closed and cannot be used anymore");
|
||||
}
|
||||
if (!zkClient.isConnected() || zkClient.isClosed()) {
|
||||
log.error("Failed to add trigger " + newTrigger.getName() + " - closing or disconnected from ZK", e);
|
||||
} else {
|
||||
log.error("Failed to add trigger " + newTrigger.getName(), e);
|
||||
}
|
||||
return;
|
||||
}
|
||||
ScheduledTrigger scheduledTrigger = st;
|
||||
|
||||
ScheduledTrigger old = scheduledTriggers.putIfAbsent(newTrigger.getName(), scheduledTrigger);
|
||||
if (old != null) {
|
||||
if (old.trigger.equals(newTrigger)) {
|
||||
|
|
|
@ -308,6 +308,8 @@ public class StreamHandler extends RequestHandlerBase implements SolrCoreAware,
|
|||
.withFunctionName("zipFDistribution", ZipFDistributionEvaluator.class)
|
||||
.withFunctionName("gammaDistribution", GammaDistributionEvaluator.class)
|
||||
.withFunctionName("betaDistribution", BetaDistributionEvaluator.class)
|
||||
.withFunctionName("polyfit", PolyFitEvaluator.class)
|
||||
.withFunctionName("polyfitDerivative", PolyFitDerivativeEvaluator.class)
|
||||
|
||||
// Boolean Stream Evaluators
|
||||
|
||||
|
|
|
@ -49,6 +49,7 @@ abstract class FacetRequestSorted extends FacetRequest {
|
|||
|
||||
|
||||
public class FacetField extends FacetRequestSorted {
|
||||
public static final int DEFAULT_FACET_LIMIT = 10;
|
||||
String field;
|
||||
boolean missing;
|
||||
boolean allBuckets; // show cumulative stats across all buckets (this can be different than non-bucketed stats across all docs because of multi-valued docs)
|
||||
|
@ -63,7 +64,7 @@ public class FacetField extends FacetRequestSorted {
|
|||
{
|
||||
// defaults for FacetRequestSorted
|
||||
mincount = 1;
|
||||
limit = 10;
|
||||
limit = DEFAULT_FACET_LIMIT;
|
||||
}
|
||||
|
||||
public enum FacetMethod {
|
||||
|
|
|
@ -44,6 +44,7 @@ import org.apache.solr.common.SolrInputDocument;
|
|||
import org.apache.solr.common.params.ModifiableSolrParams;
|
||||
import org.apache.solr.common.params.SolrParams;
|
||||
import org.apache.solr.common.util.NamedList;
|
||||
import org.apache.solr.search.facet.FacetField;
|
||||
import org.junit.AfterClass;
|
||||
import org.junit.BeforeClass;
|
||||
import org.slf4j.Logger;
|
||||
|
@ -60,15 +61,18 @@ import org.slf4j.LoggerFactory;
|
|||
* @see TestCloudPivotFacet
|
||||
*/
|
||||
public class TestCloudJSONFacetJoinDomain extends SolrCloudTestCase {
|
||||
|
||||
|
||||
private static final Logger log = LoggerFactory.getLogger(MethodHandles.lookup().lookupClass());
|
||||
|
||||
private static final String DEBUG_LABEL = MethodHandles.lookup().lookupClass().getName();
|
||||
private static final String COLLECTION_NAME = DEBUG_LABEL + "_collection";
|
||||
|
||||
private static final int DEFAULT_LIMIT = FacetField.DEFAULT_FACET_LIMIT;
|
||||
private static final int MAX_FIELD_NUM = 15;
|
||||
private static final int UNIQUE_FIELD_VALS = 20;
|
||||
private static final int FACET_LIMIT = UNIQUE_FIELD_VALS + 1;
|
||||
|
||||
// NOTE: set to 'true' to see if refinement testing is adequate (should get fails occasionally)
|
||||
private static final boolean FORCE_DISABLE_REFINEMENT = false;
|
||||
|
||||
/** Multivalued string field suffixes that can be randomized for testing diff facet/join code paths */
|
||||
private static final String[] STR_FIELD_SUFFIXES = new String[] { "_ss", "_sds", "_sdsS" };
|
||||
|
@ -88,8 +92,6 @@ public class TestCloudJSONFacetJoinDomain extends SolrCloudTestCase {
|
|||
@BeforeClass
|
||||
private static void createMiniSolrCloudCluster() throws Exception {
|
||||
// sanity check constants
|
||||
assertTrue("bad test constants: must have UNIQUE_FIELD_VALS < FACET_LIMIT to get accurate counts without refinements",
|
||||
UNIQUE_FIELD_VALS < FACET_LIMIT);
|
||||
assertTrue("bad test constants: some suffixes will never be tested",
|
||||
(STR_FIELD_SUFFIXES.length < MAX_FIELD_NUM) && (INT_FIELD_SUFFIXES.length < MAX_FIELD_NUM));
|
||||
|
||||
|
@ -170,14 +172,14 @@ public class TestCloudJSONFacetJoinDomain extends SolrCloudTestCase {
|
|||
/**
|
||||
* Given a (random) field number, returns a random (integer based) value for that field.
|
||||
* NOTE: The number of unique values in each field is constant acording to {@link #UNIQUE_FIELD_VALS}
|
||||
* but the cise pr<em>range</em> of values will vary for each unique field number, such that cross field joins
|
||||
* but the precise <em>range</em> of values will vary for each unique field number, such that cross field joins
|
||||
* will match fewer documents based on how far apart the field numbers are.
|
||||
*
|
||||
* @see #UNIQUE_FIELD_VALS
|
||||
* @see #field
|
||||
*/
|
||||
private static String randFieldValue(final int fieldNum) {
|
||||
return "" + (fieldNum + TestUtil.nextInt(random(), 0, UNIQUE_FIELD_VALS));
|
||||
return "" + (fieldNum + TestUtil.nextInt(random(), 1, UNIQUE_FIELD_VALS));
|
||||
}
|
||||
|
||||
|
||||
|
@ -301,6 +303,64 @@ public class TestCloudJSONFacetJoinDomain extends SolrCloudTestCase {
|
|||
assertFacetCountsAreCorrect(facets, "("+strfield(7)+":6 OR "+strfield(9)+":6 OR "+strfield(6)+":19 OR "+strfield(0)+":11)");
|
||||
|
||||
}
|
||||
|
||||
{ // low limits, explicit refinement
|
||||
Map<String,TermFacet> facets = new LinkedHashMap<>();
|
||||
TermFacet top = new TermFacet(strfield(9),
|
||||
new JoinDomain(strfield(5), strfield(9), strfield(9)+":[* TO *]"),
|
||||
5, 0, true);
|
||||
top.subFacets.put("facet_5", new TermFacet(strfield(11),
|
||||
new JoinDomain(strfield(8), strfield(8), null),
|
||||
10, 0, true));
|
||||
facets.put("facet_4", top);
|
||||
assertFacetCountsAreCorrect(facets, "("+strfield(7)+":6 OR "+strfield(9)+":6 OR "+strfield(6)+":19 OR "+strfield(0)+":11)");
|
||||
}
|
||||
|
||||
{ // low limit, high overrequest
|
||||
Map<String,TermFacet> facets = new LinkedHashMap<>();
|
||||
TermFacet top = new TermFacet(strfield(9),
|
||||
new JoinDomain(strfield(5), strfield(9), strfield(9)+":[* TO *]"),
|
||||
5, UNIQUE_FIELD_VALS + 10, false);
|
||||
top.subFacets.put("facet_5", new TermFacet(strfield(11),
|
||||
new JoinDomain(strfield(8), strfield(8), null),
|
||||
10, UNIQUE_FIELD_VALS + 10, false));
|
||||
facets.put("facet_4", top);
|
||||
assertFacetCountsAreCorrect(facets, "("+strfield(7)+":6 OR "+strfield(9)+":6 OR "+strfield(6)+":19 OR "+strfield(0)+":11)");
|
||||
}
|
||||
|
||||
{ // low limit, low overrequest, explicit refinement
|
||||
Map<String,TermFacet> facets = new LinkedHashMap<>();
|
||||
TermFacet top = new TermFacet(strfield(9),
|
||||
new JoinDomain(strfield(5), strfield(9), strfield(9)+":[* TO *]"),
|
||||
5, 7, true);
|
||||
top.subFacets.put("facet_5", new TermFacet(strfield(11),
|
||||
new JoinDomain(strfield(8), strfield(8), null),
|
||||
10, 7, true));
|
||||
facets.put("facet_4", top);
|
||||
assertFacetCountsAreCorrect(facets, "("+strfield(7)+":6 OR "+strfield(9)+":6 OR "+strfield(6)+":19 OR "+strfield(0)+":11)");
|
||||
}
|
||||
|
||||
}
|
||||
|
||||
public void testTheTestRandomRefineParam() {
|
||||
// sanity check that randomRefineParam never violates isRefinementNeeded
|
||||
// (should be imposisble ... unless someone changes/breaks the randomization logic in the future)
|
||||
final int numIters = atLeast(100);
|
||||
for (int iter = 0; iter < numIters; iter++) {
|
||||
final Integer limit = TermFacet.randomLimitParam(random());
|
||||
final Integer overrequest = TermFacet.randomOverrequestParam(random());
|
||||
final Boolean refine = TermFacet.randomRefineParam(random(), limit, overrequest);
|
||||
if (TermFacet.isRefinementNeeded(limit, overrequest)) {
|
||||
assertEquals("limit: " + limit + ", overrequest: " + overrequest + ", refine: " + refine,
|
||||
Boolean.TRUE, refine);
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
public void testTheTestTermFacetShouldFreakOutOnBadRefineOptions() {
|
||||
expectThrows(AssertionError.class, () -> {
|
||||
final TermFacet bogus = new TermFacet("foo", null, 5, 0, false);
|
||||
});
|
||||
}
|
||||
|
||||
public void testRandom() throws Exception {
|
||||
|
@ -423,10 +483,25 @@ public class TestCloudJSONFacetJoinDomain extends SolrCloudTestCase {
|
|||
public final String field;
|
||||
public final Map<String,TermFacet> subFacets = new LinkedHashMap<>();
|
||||
public final JoinDomain domain; // may be null
|
||||
public final Integer limit; // may be null
|
||||
public final Integer overrequest; // may be null
|
||||
public final Boolean refine; // may be null
|
||||
|
||||
/** Simplified constructor asks for limit = # unique vals */
|
||||
public TermFacet(String field, JoinDomain domain) {
|
||||
this(field, domain, UNIQUE_FIELD_VALS, 0, false);
|
||||
}
|
||||
public TermFacet(String field, JoinDomain domain, Integer limit, Integer overrequest, Boolean refine) {
|
||||
assert null != field;
|
||||
this.field = field;
|
||||
this.domain = domain;
|
||||
this.limit = limit;
|
||||
this.overrequest = overrequest;
|
||||
this.refine = refine;
|
||||
if (isRefinementNeeded(limit, overrequest)) {
|
||||
assertEquals("Invalid refine param based on limit & overrequest: " + this.toString(),
|
||||
Boolean.TRUE, refine);
|
||||
}
|
||||
}
|
||||
|
||||
/**
|
||||
|
@ -455,45 +530,10 @@ public class TestCloudJSONFacetJoinDomain extends SolrCloudTestCase {
|
|||
* recursively generates the <code>json.facet</code> param value to use for testing this facet
|
||||
*/
|
||||
private CharSequence toJSONFacetParamValue() {
|
||||
int limit = random().nextInt(FACET_LIMIT*2);
|
||||
String limitStr = ", limit:" + limit;
|
||||
if (limit >= FACET_LIMIT && random().nextBoolean()) {
|
||||
limitStr = ", limit:-1"; // unlimited
|
||||
} else if (limit == 10 && random().nextBoolean()) {
|
||||
limitStr=""; // don't specify limit since it's the default
|
||||
}
|
||||
|
||||
int overrequest = -1;
|
||||
switch(random().nextInt(10)) {
|
||||
case 0:
|
||||
case 1:
|
||||
case 2:
|
||||
case 3:
|
||||
overrequest = 0; // 40% of the time, no overrequest to better stress refinement
|
||||
break;
|
||||
case 4:
|
||||
case 5:
|
||||
overrequest = random().nextInt(FACET_LIMIT);
|
||||
break;
|
||||
case 6:
|
||||
overrequest = random().nextInt(Integer.MAX_VALUE);
|
||||
break;
|
||||
default: break;
|
||||
}
|
||||
String overrequestStr = overrequest==-1 ? "" : ", overrequest:"+overrequest;
|
||||
|
||||
boolean refine = (overrequest >= 0 && (long)limit + overrequest < FACET_LIMIT)
|
||||
|| (overrequest < 0 && limit < FACET_LIMIT) // don't assume how much overrequest we do by default, just check the limit
|
||||
|| random().nextInt(10)==0; // once in a while, turn on refinement even when it isn't needed.
|
||||
|
||||
// refine = false; // NOTE: Uncomment this line to see if refinement testing is adequate (should get fails occasionally)
|
||||
String refineStr=", refine:" + refine;
|
||||
if (!refine) {
|
||||
// if refine==false, don't specify it sometimes (it's the default)
|
||||
if (random().nextBoolean()) refineStr="";
|
||||
}
|
||||
|
||||
StringBuilder sb = new StringBuilder("{ type:terms, field:" + field + limitStr + overrequestStr + refineStr);
|
||||
final String limitStr = (null == limit) ? "" : (", limit:" + limit);
|
||||
final String overrequestStr = (null == overrequest) ? "" : (", overrequest:" + overrequest);
|
||||
final String refineStr = (null == refine) ? "" : ", refine:" + refine;
|
||||
final StringBuilder sb = new StringBuilder("{ type:terms, field:" + field + limitStr + overrequestStr + refineStr);
|
||||
if (! subFacets.isEmpty()) {
|
||||
sb.append(", facet:");
|
||||
sb.append(toJSONFacetParamValue(subFacets));
|
||||
|
@ -539,6 +579,91 @@ public class TestCloudJSONFacetJoinDomain extends SolrCloudTestCase {
|
|||
return buildRandomFacets(keyCounter, maxDepth);
|
||||
}
|
||||
|
||||
/**
|
||||
* picks a random value for the "limit" param, biased in favor of interesting test cases
|
||||
*
|
||||
* @return a number to specify in the request, or null to specify nothing (trigger default behavior)
|
||||
* @see #UNIQUE_FIELD_VALS
|
||||
*/
|
||||
public static Integer randomLimitParam(Random r) {
|
||||
final int limit = r.nextInt(UNIQUE_FIELD_VALS * 2);
|
||||
if (limit >= UNIQUE_FIELD_VALS && r.nextBoolean()) {
|
||||
return -1; // unlimited
|
||||
} else if (limit == DEFAULT_LIMIT && r.nextBoolean()) {
|
||||
return null; // sometimes, don't specify limit if it's the default
|
||||
}
|
||||
return limit;
|
||||
}
|
||||
|
||||
/**
|
||||
* picks a random value for the "overrequest" param, biased in favor of interesting test cases
|
||||
*
|
||||
* @return a number to specify in the request, or null to specify nothing (trigger default behavior)
|
||||
* @see #UNIQUE_FIELD_VALS
|
||||
*/
|
||||
public static Integer randomOverrequestParam(Random r) {
|
||||
switch(r.nextInt(10)) {
|
||||
case 0:
|
||||
case 1:
|
||||
case 2:
|
||||
case 3:
|
||||
return 0; // 40% of the time, no overrequest to better stress refinement
|
||||
case 4:
|
||||
case 5:
|
||||
return r.nextInt(UNIQUE_FIELD_VALS); // 20% ask for less them what's needed
|
||||
case 6:
|
||||
return r.nextInt(Integer.MAX_VALUE); // 10%: completley random value, statisticaly more then enough
|
||||
default: break;
|
||||
}
|
||||
// else.... either leave param unspecified (or redundently specify the -1 default)
|
||||
return r.nextBoolean() ? null : -1;
|
||||
}
|
||||
|
||||
/**
|
||||
* picks a random value for the "refine" param, that is garunteed to be suitable for
|
||||
* the specified limit & overrequest params.
|
||||
*
|
||||
* @return a value to specify in the request, or null to specify nothing (trigger default behavior)
|
||||
* @see #randomLimitParam
|
||||
* @see #randomOverrequestParam
|
||||
* @see #UNIQUE_FIELD_VALS
|
||||
*/
|
||||
public static Boolean randomRefineParam(Random r, Integer limitParam, Integer overrequestParam) {
|
||||
if (isRefinementNeeded(limitParam, overrequestParam)) {
|
||||
return true;
|
||||
}
|
||||
|
||||
// refinement is not required
|
||||
if (0 == r.nextInt(10)) { // once in a while, turn on refinement even if it isn't needed.
|
||||
return true;
|
||||
}
|
||||
// explicitly or implicitly indicate refinement is not needed
|
||||
return r.nextBoolean() ? false : null;
|
||||
}
|
||||
|
||||
/**
|
||||
* Deterministicly identifies if the specified limit & overrequest params <b>require</b>
|
||||
* a "refine:true" param be used in the the request, in order for the counts to be 100% accurate.
|
||||
*
|
||||
* @see #UNIQUE_FIELD_VALS
|
||||
*/
|
||||
public static boolean isRefinementNeeded(Integer limitParam, Integer overrequestParam) {
|
||||
|
||||
if (FORCE_DISABLE_REFINEMENT) {
|
||||
return false;
|
||||
}
|
||||
|
||||
// use the "effective" values if the params are null
|
||||
final int limit = null == limitParam ? DEFAULT_LIMIT : limitParam;
|
||||
final int overrequest = null == overrequestParam ? 0 : overrequestParam;
|
||||
|
||||
return
|
||||
// don't presume how much overrequest will be done by default, just check the limit
|
||||
(overrequest < 0 && limit < UNIQUE_FIELD_VALS)
|
||||
// if the user specified overrequest is not "enough" to get all unique values
|
||||
|| (overrequest >= 0 && (long)limit + overrequest < UNIQUE_FIELD_VALS);
|
||||
}
|
||||
|
||||
/**
|
||||
* recursive helper method for building random facets
|
||||
*
|
||||
|
@ -551,9 +676,12 @@ public class TestCloudJSONFacetJoinDomain extends SolrCloudTestCase {
|
|||
for (int i = 0; i < numFacets; i++) {
|
||||
final JoinDomain domain = JoinDomain.buildRandomDomain();
|
||||
assert null != domain;
|
||||
final Integer limit = randomLimitParam(random());
|
||||
final Integer overrequest = randomOverrequestParam(random());
|
||||
final TermFacet facet = new TermFacet(field(random().nextBoolean() ? STR_FIELD_SUFFIXES : INT_FIELD_SUFFIXES,
|
||||
random().nextInt(MAX_FIELD_NUM)),
|
||||
domain);
|
||||
domain, limit, overrequest,
|
||||
randomRefineParam(random(), limit, overrequest));
|
||||
results.put("facet_" + keyCounter.incrementAndGet(), facet);
|
||||
if (0 < maxDepth) {
|
||||
// if we're going wide, don't go deep
|
||||
|
|
|
@ -48,8 +48,8 @@ For more information about this example please read...
|
|||
|
||||
* example/solr/README.txt
|
||||
For more information about the "Solr Home" and Solr specific configuration
|
||||
* http://lucene.apache.org/solr/quickstart.html
|
||||
For a Tutorial using this example configuration
|
||||
* https://lucene.apache.org/solr/guide/solr-tutorial.html
|
||||
For a Solr tutorial
|
||||
* http://wiki.apache.org/solr/SolrResources
|
||||
For a list of other tutorials and introductory articles.
|
||||
|
||||
|
|
|
@ -23,6 +23,7 @@
|
|||
<xsl:param name="buildfiles"/>
|
||||
<xsl:param name="version"/>
|
||||
<xsl:param name="luceneJavadocUrl"/>
|
||||
<xsl:param name="solrGuideVersion"/>
|
||||
|
||||
<!--
|
||||
NOTE: This template matches the root element of any given input XML document!
|
||||
|
@ -74,7 +75,7 @@
|
|||
<li><a href="http://wiki.apache.org/solr">Wiki</a>: Additional documentation, especially focused on using Solr.</li>
|
||||
<li><a href="changes/Changes.html">Changes</a>: List of changes in this release.</li>
|
||||
<li><a href="SYSTEM_REQUIREMENTS.html">System Requirements</a>: Minimum and supported Java versions.</li>
|
||||
<li><a href="quickstart.html">Solr Quick Start</a>: This document covers the basics of running Solr using an example schema, and some sample data.</li>
|
||||
<li><a href="https://lucene.apache.org/solr/guide/{$solrGuideVersion}/solr-tutorial.html">Solr Tutorial</a>: This document covers the basics of running Solr using an example schema, and some sample data.</li>
|
||||
<li><a href="{$luceneJavadocUrl}index.html">Lucene Documentation</a></li>
|
||||
</ul>
|
||||
<h2>API Javadocs</h2>
|
||||
|
|
|
@ -1,6 +0,0 @@
|
|||
<!-- As of 2 Oct 2017, this page now automatically redirects to
|
||||
the Solr Reference Guide page solr/guide/solr-tutorial.html -->
|
||||
|
||||
# Solr Quick Start
|
||||
|
||||
Please see the [Solr Tutorial](https://lucene.apache.org/solr/guide/solr-tutorial.html) or additional [Resources](http://lucene.apache.org/solr/resources.html).
|
|
@ -113,29 +113,44 @@ public class Policy implements MapWriter {
|
|||
|
||||
}
|
||||
|
||||
private Policy(Map<String, List<Clause>> policies, List<Clause> clusterPolicy, List<Preference> clusterPreferences,
|
||||
List<String> params) {
|
||||
private Policy(Map<String, List<Clause>> policies, List<Clause> clusterPolicy, List<Preference> clusterPreferences) {
|
||||
this.policies = policies != null ? Collections.unmodifiableMap(policies) : Collections.emptyMap();
|
||||
this.clusterPolicy = clusterPolicy != null ? Collections.unmodifiableList(clusterPolicy) : Collections.emptyList();
|
||||
this.clusterPreferences = clusterPreferences != null ? Collections.unmodifiableList(clusterPreferences) :
|
||||
Collections.singletonList(DEFAULT_PREFERENCE);
|
||||
this.params = params != null ? Collections.unmodifiableList(params) : Collections.emptyList();
|
||||
this.params = Collections.unmodifiableList(buildParams(this.clusterPreferences, this.clusterPolicy, this.policies));
|
||||
}
|
||||
|
||||
private List<String> buildParams(List<Preference> preferences, List<Clause> policy, Map<String, List<Clause>> policies) {
|
||||
final SortedSet<String> paramsOfInterest = new TreeSet<>();
|
||||
preferences.forEach(p -> {
|
||||
if (paramsOfInterest.contains(p.name.name())) {
|
||||
throw new RuntimeException(p.name + " is repeated");
|
||||
}
|
||||
paramsOfInterest.add(p.name.toString());
|
||||
});
|
||||
List<String> newParams = new ArrayList<>(paramsOfInterest);
|
||||
policy.forEach(c -> {
|
||||
c.addTags(newParams);
|
||||
});
|
||||
policies.values().forEach(clauses -> clauses.forEach(c -> c.addTags(newParams)));
|
||||
return newParams;
|
||||
}
|
||||
|
||||
public Policy withPolicies(Map<String, List<Clause>> policies) {
|
||||
return new Policy(policies, clusterPolicy, clusterPreferences, params);
|
||||
return new Policy(policies, clusterPolicy, clusterPreferences);
|
||||
}
|
||||
|
||||
public Policy withClusterPreferences(List<Preference> clusterPreferences) {
|
||||
return new Policy(policies, clusterPolicy, clusterPreferences, params);
|
||||
return new Policy(policies, clusterPolicy, clusterPreferences);
|
||||
}
|
||||
|
||||
public Policy withClusterPolicy(List<Clause> clusterPolicy) {
|
||||
return new Policy(policies, clusterPolicy, clusterPreferences, params);
|
||||
return new Policy(policies, clusterPolicy, clusterPreferences);
|
||||
}
|
||||
|
||||
public Policy withParams(List<String> params) {
|
||||
return new Policy(policies, clusterPolicy, clusterPreferences, params);
|
||||
return new Policy(policies, clusterPolicy, clusterPreferences);
|
||||
}
|
||||
|
||||
public List<Clause> getClusterPolicy() {
|
||||
|
|
|
@ -0,0 +1,104 @@
|
|||
/*
|
||||
* Licensed to the Apache Software Foundation (ASF) under one or more
|
||||
* contributor license agreements. See the NOTICE file distributed with
|
||||
* this work for additional information regarding copyright ownership.
|
||||
* The ASF 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 org.apache.solr.client.solrj.io.eval;
|
||||
|
||||
import java.io.IOException;
|
||||
import java.math.BigDecimal;
|
||||
import java.util.List;
|
||||
import java.util.ArrayList;
|
||||
|
||||
import org.apache.commons.math3.analysis.UnivariateFunction;
|
||||
import org.apache.commons.math3.analysis.polynomials.PolynomialFunction;
|
||||
import org.apache.commons.math3.fitting.PolynomialCurveFitter;
|
||||
|
||||
import org.apache.commons.math3.fitting.WeightedObservedPoints;
|
||||
import org.apache.solr.client.solrj.io.stream.expr.StreamExpression;
|
||||
import org.apache.solr.client.solrj.io.stream.expr.StreamFactory;
|
||||
|
||||
public class PolyFitDerivativeEvaluator extends RecursiveNumericEvaluator implements ManyValueWorker {
|
||||
protected static final long serialVersionUID = 1L;
|
||||
|
||||
public PolyFitDerivativeEvaluator(StreamExpression expression, StreamFactory factory) throws IOException{
|
||||
super(expression, factory);
|
||||
}
|
||||
|
||||
@Override
|
||||
public Object doWork(Object... objects) throws IOException{
|
||||
|
||||
if(objects.length > 3) {
|
||||
throw new IOException("polyfitDerivative function takes a maximum of 3 arguments.");
|
||||
}
|
||||
|
||||
Object first = objects[0];
|
||||
|
||||
double[] x = null;
|
||||
double[] y = null;
|
||||
int degree = 3;
|
||||
|
||||
if(objects.length == 1) {
|
||||
//Only the y values passed
|
||||
|
||||
y = ((List) first).stream().mapToDouble(value -> ((BigDecimal) value).doubleValue()).toArray();
|
||||
x = new double[y.length];
|
||||
for(int i=0; i<y.length; i++) {
|
||||
x[i] = i;
|
||||
}
|
||||
|
||||
} else if(objects.length == 3) {
|
||||
// x, y and degree passed
|
||||
|
||||
Object second = objects[1];
|
||||
x = ((List) first).stream().mapToDouble(value -> ((BigDecimal) value).doubleValue()).toArray();
|
||||
y = ((List) second).stream().mapToDouble(value -> ((BigDecimal) value).doubleValue()).toArray();
|
||||
degree = ((Number)objects[2]).intValue();
|
||||
} else if(objects.length == 2) {
|
||||
if(objects[1] instanceof List) {
|
||||
// x and y passed
|
||||
Object second = objects[1];
|
||||
x = ((List) first).stream().mapToDouble(value -> ((BigDecimal) value).doubleValue()).toArray();
|
||||
y = ((List) second).stream().mapToDouble(value -> ((BigDecimal) value).doubleValue()).toArray();
|
||||
} else {
|
||||
// y and degree passed
|
||||
y = ((List) first).stream().mapToDouble(value -> ((BigDecimal) value).doubleValue()).toArray();
|
||||
x = new double[y.length];
|
||||
for(int i=0; i<y.length; i++) {
|
||||
x[i] = i;
|
||||
}
|
||||
|
||||
degree = ((Number)objects[1]).intValue();
|
||||
}
|
||||
}
|
||||
|
||||
PolynomialCurveFitter curveFitter = PolynomialCurveFitter.create(degree);
|
||||
WeightedObservedPoints points = new WeightedObservedPoints();
|
||||
for(int i=0; i<x.length; i++) {
|
||||
points.add(x[i], y[i]);
|
||||
}
|
||||
|
||||
double[] coef = curveFitter.fit(points.toList());
|
||||
PolynomialFunction pf = new PolynomialFunction(coef);
|
||||
UnivariateFunction univariateFunction = pf.derivative();
|
||||
|
||||
List list = new ArrayList();
|
||||
for(double xvalue : x) {
|
||||
double yvalue= univariateFunction.value(xvalue);
|
||||
list.add(yvalue);
|
||||
}
|
||||
|
||||
return list;
|
||||
}
|
||||
}
|
|
@ -0,0 +1,102 @@
|
|||
/*
|
||||
* Licensed to the Apache Software Foundation (ASF) under one or more
|
||||
* contributor license agreements. See the NOTICE file distributed with
|
||||
* this work for additional information regarding copyright ownership.
|
||||
* The ASF 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 org.apache.solr.client.solrj.io.eval;
|
||||
|
||||
import java.io.IOException;
|
||||
import java.math.BigDecimal;
|
||||
import java.util.List;
|
||||
import java.util.ArrayList;
|
||||
|
||||
import org.apache.commons.math3.analysis.polynomials.PolynomialFunction;
|
||||
import org.apache.commons.math3.fitting.PolynomialCurveFitter;
|
||||
|
||||
import org.apache.commons.math3.fitting.WeightedObservedPoints;
|
||||
import org.apache.solr.client.solrj.io.stream.expr.StreamExpression;
|
||||
import org.apache.solr.client.solrj.io.stream.expr.StreamFactory;
|
||||
|
||||
public class PolyFitEvaluator extends RecursiveNumericEvaluator implements ManyValueWorker {
|
||||
protected static final long serialVersionUID = 1L;
|
||||
|
||||
public PolyFitEvaluator(StreamExpression expression, StreamFactory factory) throws IOException{
|
||||
super(expression, factory);
|
||||
}
|
||||
|
||||
@Override
|
||||
public Object doWork(Object... objects) throws IOException{
|
||||
|
||||
if(objects.length > 3) {
|
||||
throw new IOException("polyfit function takes a maximum of 3 arguments.");
|
||||
}
|
||||
|
||||
Object first = objects[0];
|
||||
|
||||
double[] x = null;
|
||||
double[] y = null;
|
||||
int degree = 3;
|
||||
|
||||
if(objects.length == 1) {
|
||||
//Only the y values passed
|
||||
|
||||
y = ((List) first).stream().mapToDouble(value -> ((BigDecimal) value).doubleValue()).toArray();
|
||||
x = new double[y.length];
|
||||
for(int i=0; i<y.length; i++) {
|
||||
x[i] = i;
|
||||
}
|
||||
|
||||
} else if(objects.length == 3) {
|
||||
// x, y and degree passed
|
||||
|
||||
Object second = objects[1];
|
||||
x = ((List) first).stream().mapToDouble(value -> ((BigDecimal) value).doubleValue()).toArray();
|
||||
y = ((List) second).stream().mapToDouble(value -> ((BigDecimal) value).doubleValue()).toArray();
|
||||
degree = ((Number)objects[2]).intValue();
|
||||
} else if(objects.length == 2) {
|
||||
if(objects[1] instanceof List) {
|
||||
// x and y passed
|
||||
Object second = objects[1];
|
||||
x = ((List) first).stream().mapToDouble(value -> ((BigDecimal) value).doubleValue()).toArray();
|
||||
y = ((List) second).stream().mapToDouble(value -> ((BigDecimal) value).doubleValue()).toArray();
|
||||
} else {
|
||||
// y and degree passed
|
||||
y = ((List) first).stream().mapToDouble(value -> ((BigDecimal) value).doubleValue()).toArray();
|
||||
x = new double[y.length];
|
||||
for(int i=0; i<y.length; i++) {
|
||||
x[i] = i;
|
||||
}
|
||||
|
||||
degree = ((Number)objects[1]).intValue();
|
||||
}
|
||||
}
|
||||
|
||||
PolynomialCurveFitter curveFitter = PolynomialCurveFitter.create(degree);
|
||||
WeightedObservedPoints points = new WeightedObservedPoints();
|
||||
for(int i=0; i<x.length; i++) {
|
||||
points.add(x[i], y[i]);
|
||||
}
|
||||
|
||||
double[] coef = curveFitter.fit(points.toList());
|
||||
PolynomialFunction pf = new PolynomialFunction(coef);
|
||||
|
||||
List list = new ArrayList();
|
||||
for(double xvalue : x) {
|
||||
double yvalue= pf.value(xvalue);
|
||||
list.add(yvalue);
|
||||
}
|
||||
|
||||
return list;
|
||||
}
|
||||
}
|
|
@ -20,7 +20,6 @@ package org.apache.solr.common;
|
|||
|
||||
import java.io.IOException;
|
||||
import java.util.ArrayList;
|
||||
import java.util.HashMap;
|
||||
import java.util.LinkedHashMap;
|
||||
import java.util.List;
|
||||
import java.util.Map;
|
||||
|
|
|
@ -35,7 +35,6 @@ import org.apache.solr.client.solrj.SolrRequest;
|
|||
import org.apache.solr.client.solrj.cloud.autoscaling.Clause.Violation;
|
||||
import org.apache.solr.client.solrj.cloud.autoscaling.Policy.Suggester.Hint;
|
||||
import org.apache.solr.client.solrj.request.CollectionAdminRequest;
|
||||
import org.apache.solr.cloud.autoscaling.TriggerEvent;
|
||||
import org.apache.solr.common.cloud.Replica;
|
||||
import org.apache.solr.common.cloud.ReplicaPosition;
|
||||
import org.apache.solr.common.cloud.ZkStateReader;
|
||||
|
|
|
@ -6541,6 +6541,44 @@ public class StreamExpressionTest extends SolrCloudTestCase {
|
|||
assertTrue(out.get(5).intValue() == -6);
|
||||
}
|
||||
|
||||
@Test
|
||||
public void testPolyfit() throws Exception {
|
||||
String cexpr = "let(echo=true," +
|
||||
" a=array(0,1,2,3,4,5,6,7)," +
|
||||
" fit=polyfit(a, 1)," +
|
||||
" deriv=polyfitDerivative(a, 1))";
|
||||
ModifiableSolrParams paramsLoc = new ModifiableSolrParams();
|
||||
paramsLoc.set("expr", cexpr);
|
||||
paramsLoc.set("qt", "/stream");
|
||||
String url = cluster.getJettySolrRunners().get(0).getBaseUrl().toString()+"/"+COLLECTIONORALIAS;
|
||||
TupleStream solrStream = new SolrStream(url, paramsLoc);
|
||||
StreamContext context = new StreamContext();
|
||||
solrStream.setStreamContext(context);
|
||||
List<Tuple> tuples = getTuples(solrStream);
|
||||
assertTrue(tuples.size() == 1);
|
||||
List<Number> out = (List<Number>)tuples.get(0).get("fit");
|
||||
assertTrue(out.size() == 8);
|
||||
assertTrue(out.get(0).intValue() == 0);
|
||||
assertTrue(out.get(1).intValue() == 1);
|
||||
assertTrue(out.get(2).intValue() == 2);
|
||||
assertTrue(out.get(3).intValue() == 3);
|
||||
assertTrue(out.get(4).intValue() == 4);
|
||||
assertTrue(out.get(5).intValue() == 5);
|
||||
assertTrue(out.get(6).intValue() == 6);
|
||||
assertTrue(out.get(7).intValue() == 7);
|
||||
|
||||
out = (List<Number>)tuples.get(0).get("deriv");
|
||||
assertTrue(out.size() == 8);
|
||||
assertTrue(out.get(0).intValue() == 1);
|
||||
assertTrue(out.get(1).intValue() == 1);
|
||||
assertTrue(out.get(2).intValue() == 1);
|
||||
assertTrue(out.get(3).intValue() == 1);
|
||||
assertTrue(out.get(4).intValue() == 1);
|
||||
assertTrue(out.get(5).intValue() == 1);
|
||||
assertTrue(out.get(6).intValue() == 1);
|
||||
assertTrue(out.get(7).intValue() == 1);
|
||||
}
|
||||
|
||||
|
||||
@Test
|
||||
public void testAnova() throws Exception {
|
||||
|
@ -7518,14 +7556,14 @@ public class StreamExpressionTest extends SolrCloudTestCase {
|
|||
|
||||
@Test
|
||||
public void testParallelExecutorStream() throws Exception {
|
||||
CollectionAdminRequest.createCollection("workQueue", "conf", 2, 1).processAndWait(cluster.getSolrClient(),DEFAULT_TIMEOUT);
|
||||
AbstractDistribZkTestBase.waitForRecoveriesToFinish("workQueue", cluster.getSolrClient().getZkStateReader(),
|
||||
CollectionAdminRequest.createCollection("workQueue1", "conf", 2, 1).processAndWait(cluster.getSolrClient(),DEFAULT_TIMEOUT);
|
||||
AbstractDistribZkTestBase.waitForRecoveriesToFinish("workQueue1", cluster.getSolrClient().getZkStateReader(),
|
||||
false, true, TIMEOUT);
|
||||
CollectionAdminRequest.createCollection("mainCorpus", "conf", 2, 1).processAndWait(cluster.getSolrClient(), DEFAULT_TIMEOUT);
|
||||
AbstractDistribZkTestBase.waitForRecoveriesToFinish("mainCorpus", cluster.getSolrClient().getZkStateReader(),
|
||||
CollectionAdminRequest.createCollection("mainCorpus1", "conf", 2, 1).processAndWait(cluster.getSolrClient(), DEFAULT_TIMEOUT);
|
||||
AbstractDistribZkTestBase.waitForRecoveriesToFinish("mainCorpus1", cluster.getSolrClient().getZkStateReader(),
|
||||
false, true, TIMEOUT);
|
||||
CollectionAdminRequest.createCollection("destination", "conf", 2, 1).processAndWait(cluster.getSolrClient(), DEFAULT_TIMEOUT);
|
||||
AbstractDistribZkTestBase.waitForRecoveriesToFinish("destination", cluster.getSolrClient().getZkStateReader(),
|
||||
CollectionAdminRequest.createCollection("destination1", "conf", 2, 1).processAndWait(cluster.getSolrClient(), DEFAULT_TIMEOUT);
|
||||
AbstractDistribZkTestBase.waitForRecoveriesToFinish("destination1", cluster.getSolrClient().getZkStateReader(),
|
||||
false, true, TIMEOUT);
|
||||
|
||||
UpdateRequest workRequest = new UpdateRequest();
|
||||
|
@ -7533,27 +7571,27 @@ public class StreamExpressionTest extends SolrCloudTestCase {
|
|||
|
||||
|
||||
for (int i = 0; i < 500; i++) {
|
||||
workRequest.add(id, String.valueOf(i), "expr_s", "update(destination, batchSize=50, search(mainCorpus, q=id:"+i+", rows=1, sort=\"id asc\", fl=\"id, body_t, field_i\"))");
|
||||
workRequest.add(id, String.valueOf(i), "expr_s", "update(destination1, batchSize=50, search(mainCorpus1, q=id:"+i+", rows=1, sort=\"id asc\", fl=\"id, body_t, field_i\"))");
|
||||
dataRequest.add(id, String.valueOf(i), "body_t", "hello world "+i, "field_i", Integer.toString(i));
|
||||
}
|
||||
|
||||
workRequest.commit(cluster.getSolrClient(), "workQueue");
|
||||
dataRequest.commit(cluster.getSolrClient(), "mainCorpus");
|
||||
workRequest.commit(cluster.getSolrClient(), "workQueue1");
|
||||
dataRequest.commit(cluster.getSolrClient(), "mainCorpus1");
|
||||
|
||||
String url = cluster.getJettySolrRunners().get(0).getBaseUrl().toString() + "/destination";
|
||||
String url = cluster.getJettySolrRunners().get(0).getBaseUrl().toString() + "/destination1";
|
||||
TupleStream executorStream;
|
||||
ModifiableSolrParams paramsLoc;
|
||||
|
||||
StreamFactory factory = new StreamFactory()
|
||||
.withCollectionZkHost("workQueue", cluster.getZkServer().getZkAddress())
|
||||
.withCollectionZkHost("mainCorpus", cluster.getZkServer().getZkAddress())
|
||||
.withCollectionZkHost("destination", cluster.getZkServer().getZkAddress())
|
||||
.withCollectionZkHost("workQueue1", cluster.getZkServer().getZkAddress())
|
||||
.withCollectionZkHost("mainCorpus1", cluster.getZkServer().getZkAddress())
|
||||
.withCollectionZkHost("destination1", cluster.getZkServer().getZkAddress())
|
||||
.withFunctionName("search", CloudSolrStream.class)
|
||||
.withFunctionName("executor", ExecutorStream.class)
|
||||
.withFunctionName("parallel", ParallelStream.class)
|
||||
.withFunctionName("update", UpdateStream.class);
|
||||
|
||||
String executorExpression = "parallel(workQueue, workers=2, sort=\"EOF asc\", executor(threads=3, queueSize=100, search(workQueue, q=\"*:*\", fl=\"id, expr_s\", rows=1000, partitionKeys=id, sort=\"id desc\")))";
|
||||
String executorExpression = "parallel(workQueue1, workers=2, sort=\"EOF asc\", executor(threads=3, queueSize=100, search(workQueue1, q=\"*:*\", fl=\"id, expr_s\", rows=1000, partitionKeys=id, sort=\"id desc\")))";
|
||||
executorStream = factory.constructStream(executorExpression);
|
||||
|
||||
StreamContext context = new StreamContext();
|
||||
|
@ -7562,9 +7600,9 @@ public class StreamExpressionTest extends SolrCloudTestCase {
|
|||
executorStream.setStreamContext(context);
|
||||
getTuples(executorStream);
|
||||
//Destination collection should now contain all the records in the main corpus.
|
||||
cluster.getSolrClient().commit("destination");
|
||||
cluster.getSolrClient().commit("destination1");
|
||||
paramsLoc = new ModifiableSolrParams();
|
||||
paramsLoc.set("expr", "search(destination, q=\"*:*\", fl=\"id, body_t, field_i\", rows=1000, sort=\"field_i asc\")");
|
||||
paramsLoc.set("expr", "search(destination1, q=\"*:*\", fl=\"id, body_t, field_i\", rows=1000, sort=\"field_i asc\")");
|
||||
paramsLoc.set("qt", "/stream");
|
||||
|
||||
SolrStream solrStream = new SolrStream(url, paramsLoc);
|
||||
|
@ -7580,9 +7618,9 @@ public class StreamExpressionTest extends SolrCloudTestCase {
|
|||
|
||||
solrStream.close();
|
||||
clientCache.close();
|
||||
CollectionAdminRequest.deleteCollection("workQueue").process(cluster.getSolrClient());
|
||||
CollectionAdminRequest.deleteCollection("mainCorpus").process(cluster.getSolrClient());
|
||||
CollectionAdminRequest.deleteCollection("destination").process(cluster.getSolrClient());
|
||||
CollectionAdminRequest.deleteCollection("workQueue1").process(cluster.getSolrClient());
|
||||
CollectionAdminRequest.deleteCollection("mainCorpus1").process(cluster.getSolrClient());
|
||||
CollectionAdminRequest.deleteCollection("destination1").process(cluster.getSolrClient());
|
||||
}
|
||||
|
||||
|
||||
|
|
Loading…
Reference in New Issue