Search: Terms Stats Facet, closes #705.

This commit is contained in:
kimchy 2011-02-19 23:37:04 +02:00
parent 745614f53d
commit 352cb74f96
14 changed files with 2212 additions and 1 deletions

View File

@ -32,6 +32,7 @@ import org.elasticsearch.search.facet.range.RangeScriptFacetBuilder;
import org.elasticsearch.search.facet.statistical.StatisticalFacetBuilder;
import org.elasticsearch.search.facet.statistical.StatisticalScriptFacetBuilder;
import org.elasticsearch.search.facet.terms.TermsFacetBuilder;
import org.elasticsearch.search.facet.termsstats.TermsStatsFacetBuilder;
/**
* @author kimchy (shay.banon)
@ -58,6 +59,10 @@ public class FacetBuilders {
return new TermsFacetBuilder(facetName);
}
public static TermsStatsFacetBuilder termsStats(String facetName) {
return new TermsStatsFacetBuilder(facetName);
}
public static StatisticalFacetBuilder statisticalFacet(String facetName) {
return new StatisticalFacetBuilder(facetName);
}

View File

@ -30,6 +30,7 @@ import org.elasticsearch.search.facet.query.QueryFacetProcessor;
import org.elasticsearch.search.facet.range.RangeFacetProcessor;
import org.elasticsearch.search.facet.statistical.StatisticalFacetProcessor;
import org.elasticsearch.search.facet.terms.TermsFacetProcessor;
import org.elasticsearch.search.facet.termsstats.TermsStatsFacetProcessor;
import java.util.List;
@ -49,6 +50,7 @@ public class FacetModule extends AbstractModule {
processors.add(RangeFacetProcessor.class);
processors.add(StatisticalFacetProcessor.class);
processors.add(TermsFacetProcessor.class);
processors.add(TermsStatsFacetProcessor.class);
}
public void addFacetProcessor(Class<? extends FacetProcessor> facetProcessor) {

View File

@ -28,6 +28,7 @@ import org.elasticsearch.search.facet.query.InternalQueryFacet;
import org.elasticsearch.search.facet.range.InternalRangeFacet;
import org.elasticsearch.search.facet.statistical.InternalStatisticalFacet;
import org.elasticsearch.search.facet.terms.InternalTermsFacet;
import org.elasticsearch.search.facet.termsstats.InternalTermsStatsFacet;
/**
* @author kimchy (shay.banon)
@ -43,5 +44,6 @@ public class TransportFacetModule extends AbstractModule {
InternalRangeFacet.registerStreams();
InternalStatisticalFacet.registerStreams();
InternalTermsFacet.registerStreams();
InternalTermsStatsFacet.registerStreams();
}
}

View File

@ -0,0 +1,39 @@
/*
* Licensed to Elastic Search and Shay Banon under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. Elastic Search 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.elasticsearch.search.facet.termsstats;
import org.elasticsearch.search.facet.Facet;
import org.elasticsearch.search.facet.InternalFacet;
import org.elasticsearch.search.facet.termsstats.doubles.InternalTermsStatsDoubleFacet;
import org.elasticsearch.search.facet.termsstats.longs.InternalTermsStatsLongFacet;
import org.elasticsearch.search.facet.termsstats.strings.InternalTermsStatsStringFacet;
import java.util.List;
public abstract class InternalTermsStatsFacet implements TermsStatsFacet, InternalFacet {
public static void registerStreams() {
InternalTermsStatsStringFacet.registerStream();
InternalTermsStatsLongFacet.registerStream();
InternalTermsStatsDoubleFacet.registerStream();
}
public abstract Facet reduce(String name, List<Facet> facets);
}

View File

@ -0,0 +1,187 @@
/*
* Licensed to Elastic Search and Shay Banon under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. Elastic Search 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.elasticsearch.search.facet.termsstats;
import org.elasticsearch.ElasticSearchIllegalArgumentException;
import org.elasticsearch.search.facet.Facet;
import java.util.Comparator;
import java.util.List;
public interface TermsStatsFacet extends Facet, Iterable<TermsStatsFacet.Entry> {
public static final String TYPE = "terms_stats";
/**
* The number of docs missing a value.
*/
long missingCount();
/**
* The number of docs missing a value.
*/
long getMissingCount();
/**
* The terms and counts.
*/
List<? extends TermsStatsFacet.Entry> entries();
/**
* The terms and counts.
*/
List<? extends TermsStatsFacet.Entry> getEntries();
/**
* Controls how the terms facets are ordered.
*/
public static enum ComparatorType {
/**
* Order by the (higher) count of each term.
*/
COUNT((byte) 0, new Comparator<Entry>() {
@Override public int compare(Entry o1, Entry o2) {
int i = o2.count() - o1.count();
if (i == 0) {
i = o2.term().compareTo(o1.term());
if (i == 0) {
i = System.identityHashCode(o2) - System.identityHashCode(o1);
}
}
return i;
}
}),
/**
* Order by the (lower) count of each term.
*/
REVERSE_COUNT((byte) 1, new Comparator<Entry>() {
@Override public int compare(Entry o1, Entry o2) {
return -COUNT.comparator().compare(o1, o2);
}
}),
/**
* Order by the terms.
*/
TERM((byte) 2, new Comparator<Entry>() {
@Override public int compare(Entry o1, Entry o2) {
return o1.compareTo(o2);
}
}),
/**
* Order by the terms.
*/
REVERSE_TERM((byte) 3, new Comparator<Entry>() {
@Override public int compare(Entry o1, Entry o2) {
return -TERM.comparator().compare(o1, o2);
}
}),
TOTAL((byte) 4, new Comparator<Entry>() {
@Override public int compare(Entry o1, Entry o2) {
return (o2.total() < o1.total() ? -1 : (o2.total() == o1.total() ? 0 : 1));
}
}),
REVERSE_TOTAL((byte) 5, new Comparator<Entry>() {
@Override public int compare(Entry o1, Entry o2) {
return (o1.total() < o2.total() ? -1 : (o1.total() == o2.total() ? 0 : 1));
}
});
private final byte id;
private final Comparator<Entry> comparator;
ComparatorType(byte id, Comparator<Entry> comparator) {
this.id = id;
this.comparator = comparator;
}
public byte id() {
return this.id;
}
public Comparator<Entry> comparator() {
return comparator;
}
public static ComparatorType fromId(byte id) {
if (id == COUNT.id()) {
return COUNT;
} else if (id == REVERSE_COUNT.id()) {
return REVERSE_COUNT;
} else if (id == TERM.id()) {
return TERM;
} else if (id == REVERSE_TERM.id()) {
return REVERSE_TERM;
} else if (id == TOTAL.id()) {
return TOTAL;
} else if (id == REVERSE_TOTAL.id()) {
return REVERSE_TOTAL;
}
throw new ElasticSearchIllegalArgumentException("No type argument match for terms facet comparator [" + id + "]");
}
public static ComparatorType fromString(String type) {
if ("count".equals(type)) {
return COUNT;
} else if ("term".equals(type)) {
return TERM;
} else if ("reverse_count".equals(type) || "reverseCount".equals(type)) {
return REVERSE_COUNT;
} else if ("reverse_term".equals(type) || "reverseTerm".equals(type)) {
return REVERSE_TERM;
} else if ("total".equals(type)) {
return TOTAL;
} else if ("reverse_total".equals(type) || "reverseTotal".equals(type)) {
return REVERSE_TOTAL;
}
throw new ElasticSearchIllegalArgumentException("No type argument match for terms stats facet comparator [" + type + "]");
}
}
public interface Entry extends Comparable<Entry> {
String term();
String getTerm();
Number termAsNumber();
Number getTermAsNumber();
int count();
int getCount();
double total();
double getTotal();
double mean();
double getMean();
}
}

View File

@ -0,0 +1,154 @@
/*
* Licensed to Elastic Search and Shay Banon under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. Elastic Search 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.elasticsearch.search.facet.termsstats;
import org.elasticsearch.common.collect.Maps;
import org.elasticsearch.common.xcontent.XContentBuilder;
import org.elasticsearch.search.builder.SearchSourceBuilderException;
import org.elasticsearch.search.facet.AbstractFacetBuilder;
import java.io.IOException;
import java.util.Map;
/**
*
*/
public class TermsStatsFacetBuilder extends AbstractFacetBuilder {
private String keyField;
private String valueField;
private int size = -1;
private TermsStatsFacet.ComparatorType comparatorType;
private String script;
private String lang;
private Map<String, Object> params;
/**
* Constructs a new terms stats facet builder under the provided facet name.
*/
public TermsStatsFacetBuilder(String name) {
super(name);
}
public TermsStatsFacetBuilder keyField(String keyField) {
this.keyField = keyField;
return this;
}
public TermsStatsFacetBuilder valueField(String valueField) {
this.valueField = valueField;
return this;
}
/**
* The order by which to return the facets by. Defaults to {@link TermsStatsFacet.ComparatorType#COUNT}.
*/
public TermsStatsFacetBuilder order(TermsStatsFacet.ComparatorType comparatorType) {
this.comparatorType = comparatorType;
return this;
}
/**
* Sets the size of the result.
*/
public TermsStatsFacetBuilder size(int size) {
this.size = size;
return this;
}
/**
* Marks all terms to be returned, even ones with 0 counts.
*/
public TermsStatsFacetBuilder allTerms() {
this.size = 0;
return this;
}
/**
* A value script to be executed (instead of value field) which results (numeric) will be used
* to compute the totals.
*/
public TermsStatsFacetBuilder valueScript(String script) {
this.script = script;
return this;
}
/**
* The language of the script.
*/
public TermsStatsFacetBuilder lang(String lang) {
this.lang = lang;
return this;
}
/**
* A parameter that will be passed to the script.
*
* @param name The name of the script parameter.
* @param value The value of the script parameter.
*/
public TermsStatsFacetBuilder param(String name, Object value) {
if (params == null) {
params = Maps.newHashMap();
}
params.put(name, value);
return this;
}
@Override public XContentBuilder toXContent(XContentBuilder builder, Params params) throws IOException {
if (keyField == null) {
throw new SearchSourceBuilderException("key field must be set on terms facet for facet [" + name + "]");
}
if (valueField == null && script == null) {
throw new SearchSourceBuilderException("value field or value script must be set on terms facet for facet [" + name + "]");
}
builder.startObject(name);
builder.startObject(TermsStatsFacet.TYPE);
builder.field("key_field", keyField);
if (valueField != null) {
builder.field("value_field", valueField);
}
if (script != null) {
builder.field("value_script", script);
if (lang != null) {
builder.field("lang", lang);
}
if (this.params != null) {
builder.field("params", this.params);
}
}
if (comparatorType != null) {
builder.field("order", comparatorType.name().toLowerCase());
}
if (size != -1) {
builder.field("size", size);
}
builder.endObject();
builder.endObject();
return builder;
}
}

View File

@ -0,0 +1,124 @@
/*
* Licensed to Elastic Search and Shay Banon under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. Elastic Search 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.elasticsearch.search.facet.termsstats;
import org.elasticsearch.common.component.AbstractComponent;
import org.elasticsearch.common.inject.Inject;
import org.elasticsearch.common.settings.Settings;
import org.elasticsearch.common.xcontent.XContentParser;
import org.elasticsearch.index.field.data.FieldDataType;
import org.elasticsearch.index.mapper.FieldMapper;
import org.elasticsearch.search.facet.Facet;
import org.elasticsearch.search.facet.FacetCollector;
import org.elasticsearch.search.facet.FacetPhaseExecutionException;
import org.elasticsearch.search.facet.FacetProcessor;
import org.elasticsearch.search.facet.termsstats.doubles.TermsStatsDoubleFacetCollector;
import org.elasticsearch.search.facet.termsstats.longs.TermsStatsLongFacetCollector;
import org.elasticsearch.search.facet.termsstats.strings.TermsStatsStringFacetCollector;
import org.elasticsearch.search.internal.SearchContext;
import java.io.IOException;
import java.util.List;
import java.util.Map;
public class TermsStatsFacetProcessor extends AbstractComponent implements FacetProcessor {
@Inject public TermsStatsFacetProcessor(Settings settings) {
super(settings);
InternalTermsStatsFacet.registerStreams();
}
@Override public String[] types() {
return new String[]{TermsStatsFacet.TYPE, "termsStats"};
}
@Override public FacetCollector parse(String facetName, XContentParser parser, SearchContext context) throws IOException {
String keyField = null;
String valueField = null;
int size = 10;
TermsStatsFacet.ComparatorType comparatorType = TermsStatsFacet.ComparatorType.COUNT;
String scriptLang = null;
String script = null;
Map<String, Object> params = null;
String currentFieldName = null;
XContentParser.Token token;
while ((token = parser.nextToken()) != XContentParser.Token.END_OBJECT) {
if (token == XContentParser.Token.FIELD_NAME) {
currentFieldName = parser.currentName();
} else if (token == XContentParser.Token.START_OBJECT) {
if ("params".equals(currentFieldName)) {
params = parser.map();
}
} else if (token.isValue()) {
if ("key_field".equals(currentFieldName) || "keyField".equals(currentFieldName)) {
keyField = parser.text();
} else if ("value_field".equals(currentFieldName) || "valueField".equals(currentFieldName)) {
valueField = parser.text();
} else if ("script_field".equals(currentFieldName)) {
script = parser.text();
} else if ("size".equals(currentFieldName)) {
size = parser.intValue();
} else if ("all_terms".equals(currentFieldName) || "allTerms".equals(currentFieldName)) {
if (parser.booleanValue()) {
size = 0; // indicates all terms
}
} else if ("order".equals(currentFieldName) || "comparator".equals(currentFieldName)) {
comparatorType = TermsStatsFacet.ComparatorType.fromString(parser.text());
} else if ("value_script".equals(currentFieldName)) {
script = parser.text();
} else if ("lang".equals(currentFieldName)) {
scriptLang = parser.text();
}
}
}
if (keyField == null) {
throw new FacetPhaseExecutionException(facetName, "[key_field] is required to be set for terms stats facet");
}
if (valueField == null && script == null) {
throw new FacetPhaseExecutionException(facetName, "either [value_field] or [script] are required to be set for terms stats facet");
}
FieldMapper keyFieldMapper = context.mapperService().smartNameFieldMapper(keyField);
if (keyFieldMapper != null) {
if (keyFieldMapper.fieldDataType() == FieldDataType.DefaultTypes.LONG) {
return new TermsStatsLongFacetCollector(facetName, keyField, valueField, size, comparatorType, context, scriptLang, script, params);
} else if (keyFieldMapper.fieldDataType() == FieldDataType.DefaultTypes.INT) {
return new TermsStatsLongFacetCollector(facetName, keyField, valueField, size, comparatorType, context, scriptLang, script, params);
} else if (keyFieldMapper.fieldDataType() == FieldDataType.DefaultTypes.SHORT) {
return new TermsStatsLongFacetCollector(facetName, keyField, valueField, size, comparatorType, context, scriptLang, script, params);
} else if (keyFieldMapper.fieldDataType() == FieldDataType.DefaultTypes.BYTE) {
return new TermsStatsLongFacetCollector(facetName, keyField, valueField, size, comparatorType, context, scriptLang, script, params);
} else if (keyFieldMapper.fieldDataType() == FieldDataType.DefaultTypes.DOUBLE) {
return new TermsStatsDoubleFacetCollector(facetName, keyField, valueField, size, comparatorType, context, scriptLang, script, params);
} else if (keyFieldMapper.fieldDataType() == FieldDataType.DefaultTypes.FLOAT) {
return new TermsStatsDoubleFacetCollector(facetName, keyField, valueField, size, comparatorType, context, scriptLang, script, params);
}
}
return new TermsStatsStringFacetCollector(facetName, keyField, valueField, size, comparatorType, context, scriptLang, script, params);
}
@Override public Facet reduce(String name, List<Facet> facets) {
InternalTermsStatsFacet first = (InternalTermsStatsFacet) facets.get(0);
return first.reduce(name, facets);
}
}

View File

@ -0,0 +1,285 @@
/*
* Licensed to Elastic Search and Shay Banon under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. Elastic Search 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.elasticsearch.search.facet.termsstats.doubles;
import org.elasticsearch.common.collect.BoundedTreeSet;
import org.elasticsearch.common.collect.ImmutableList;
import org.elasticsearch.common.io.stream.StreamInput;
import org.elasticsearch.common.io.stream.StreamOutput;
import org.elasticsearch.common.thread.ThreadLocals;
import org.elasticsearch.common.trove.map.hash.THashMap;
import org.elasticsearch.common.xcontent.XContentBuilder;
import org.elasticsearch.common.xcontent.XContentBuilderString;
import org.elasticsearch.search.facet.Facet;
import org.elasticsearch.search.facet.termsstats.InternalTermsStatsFacet;
import java.io.IOException;
import java.util.*;
public class InternalTermsStatsDoubleFacet extends InternalTermsStatsFacet {
private static final String STREAM_TYPE = "dTS";
public static void registerStream() {
Streams.registerStream(STREAM, STREAM_TYPE);
}
static Stream STREAM = new Stream() {
@Override public Facet readFacet(String type, StreamInput in) throws IOException {
return readTermsStatsFacet(in);
}
};
@Override public String streamType() {
return STREAM_TYPE;
}
public InternalTermsStatsDoubleFacet() {
}
public static class DoubleEntry implements Entry {
double term;
int count;
double total;
public DoubleEntry(double term, int count, double total) {
this.term = term;
this.count = count;
this.total = total;
}
@Override public String term() {
return Double.toString(term);
}
@Override public String getTerm() {
return term();
}
@Override public Number termAsNumber() {
return term;
}
@Override public Number getTermAsNumber() {
return termAsNumber();
}
@Override public int count() {
return count;
}
@Override public int getCount() {
return count();
}
@Override public double total() {
return total;
}
@Override public double getTotal() {
return total();
}
@Override public double mean() {
return total / count;
}
@Override public double getMean() {
return mean();
}
@Override public int compareTo(Entry o) {
DoubleEntry other = (DoubleEntry) o;
int i = (term < other.term ? -1 : (term == other.term ? 0 : 1));
if (i == 0) {
i = count - o.count();
if (i == 0) {
i = System.identityHashCode(this) - System.identityHashCode(o);
}
}
return i;
}
}
private String name;
int requiredSize;
long missing;
Collection<DoubleEntry> entries = ImmutableList.of();
ComparatorType comparatorType;
public InternalTermsStatsDoubleFacet(String name, ComparatorType comparatorType, int requiredSize, Collection<DoubleEntry> entries, long missing) {
this.name = name;
this.comparatorType = comparatorType;
this.requiredSize = requiredSize;
this.entries = entries;
this.missing = missing;
}
@Override public String name() {
return this.name;
}
@Override public String getName() {
return this.name;
}
@Override public String type() {
return TYPE;
}
@Override public String getType() {
return type();
}
@Override public List<DoubleEntry> entries() {
if (!(entries instanceof List)) {
entries = ImmutableList.copyOf(entries);
}
return (List<DoubleEntry>) entries;
}
@Override public List<DoubleEntry> getEntries() {
return entries();
}
@SuppressWarnings({"unchecked"}) @Override public Iterator<Entry> iterator() {
return (Iterator) entries.iterator();
}
@Override public long missingCount() {
return this.missing;
}
@Override public long getMissingCount() {
return missingCount();
}
private static ThreadLocal<ThreadLocals.CleanableValue<THashMap<String, DoubleEntry>>> aggregateCache = new ThreadLocal<ThreadLocals.CleanableValue<THashMap<String, DoubleEntry>>>() {
@Override protected ThreadLocals.CleanableValue<THashMap<String, DoubleEntry>> initialValue() {
return new ThreadLocals.CleanableValue<THashMap<String, DoubleEntry>>(new THashMap<String, DoubleEntry>());
}
};
@Override public Facet reduce(String name, List<Facet> facets) {
if (facets.size() == 1) {
if (requiredSize == 0) {
// we need to sort it here!
InternalTermsStatsDoubleFacet tsFacet = (InternalTermsStatsDoubleFacet) facets.get(0);
List<DoubleEntry> entries = tsFacet.entries();
Collections.sort(entries, comparatorType.comparator());
}
return facets.get(0);
}
int missing = 0;
THashMap<String, DoubleEntry> map = aggregateCache.get().get();
map.clear();
for (Facet facet : facets) {
InternalTermsStatsDoubleFacet tsFacet = (InternalTermsStatsDoubleFacet) facet;
missing += tsFacet.missing;
for (Entry entry : tsFacet) {
DoubleEntry doubleEntry = (DoubleEntry) entry;
DoubleEntry current = map.get(doubleEntry.term());
if (current != null) {
current.count += doubleEntry.count;
current.total += doubleEntry.total;
} else {
map.put(doubleEntry.term(), doubleEntry);
}
}
}
// sort
if (requiredSize == 0) { // all terms
DoubleEntry[] entries1 = map.values().toArray(new DoubleEntry[map.size()]);
Arrays.sort(entries1, comparatorType.comparator());
return new InternalTermsStatsDoubleFacet(name, comparatorType, requiredSize, Arrays.asList(entries1), missing);
} else {
TreeSet<DoubleEntry> ordered = new BoundedTreeSet<DoubleEntry>(comparatorType.comparator(), requiredSize);
ordered.addAll(map.values());
return new InternalTermsStatsDoubleFacet(name, comparatorType, requiredSize, ordered, missing);
}
}
static final class Fields {
static final XContentBuilderString _TYPE = new XContentBuilderString("_type");
static final XContentBuilderString MISSING = new XContentBuilderString("missing");
static final XContentBuilderString TERMS = new XContentBuilderString("terms");
static final XContentBuilderString TERM = new XContentBuilderString("term");
static final XContentBuilderString COUNT = new XContentBuilderString("count");
static final XContentBuilderString TOTAL = new XContentBuilderString("total");
static final XContentBuilderString MEAN = new XContentBuilderString("mean");
}
@Override public XContentBuilder toXContent(XContentBuilder builder, Params params) throws IOException {
builder.startObject(name);
builder.field(Fields._TYPE, InternalTermsStatsFacet.TYPE);
builder.field(Fields.MISSING, missing);
builder.startArray(Fields.TERMS);
for (Entry entry : entries) {
builder.startObject();
builder.field(Fields.TERM, ((DoubleEntry) entry).term);
builder.field(Fields.COUNT, entry.count());
builder.field(Fields.TOTAL, entry.total());
builder.field(Fields.MEAN, entry.mean());
builder.endObject();
}
builder.endArray();
builder.endObject();
return builder;
}
public static InternalTermsStatsDoubleFacet readTermsStatsFacet(StreamInput in) throws IOException {
InternalTermsStatsDoubleFacet facet = new InternalTermsStatsDoubleFacet();
facet.readFrom(in);
return facet;
}
@Override public void readFrom(StreamInput in) throws IOException {
name = in.readUTF();
comparatorType = ComparatorType.fromId(in.readByte());
requiredSize = in.readVInt();
missing = in.readVLong();
int size = in.readVInt();
entries = new ArrayList<DoubleEntry>(size);
for (int i = 0; i < size; i++) {
entries.add(new DoubleEntry(in.readDouble(), in.readVInt(), in.readDouble()));
}
}
@Override public void writeTo(StreamOutput out) throws IOException {
out.writeUTF(name);
out.writeByte(comparatorType.id());
out.writeVInt(requiredSize);
out.writeVLong(missing);
out.writeVInt(entries.size());
for (Entry entry : entries) {
out.writeDouble(((DoubleEntry) entry).term);
out.writeVInt(entry.count());
out.writeDouble(entry.total());
}
}
}

View File

@ -0,0 +1,200 @@
/*
* Licensed to Elastic Search and Shay Banon under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. Elastic Search 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.elasticsearch.search.facet.termsstats.doubles;
import org.apache.lucene.index.IndexReader;
import org.apache.lucene.search.Scorer;
import org.elasticsearch.ElasticSearchIllegalArgumentException;
import org.elasticsearch.common.collect.BoundedTreeSet;
import org.elasticsearch.common.collect.ImmutableList;
import org.elasticsearch.common.thread.ThreadLocals;
import org.elasticsearch.common.trove.map.hash.TDoubleObjectHashMap;
import org.elasticsearch.index.cache.field.data.FieldDataCache;
import org.elasticsearch.index.field.data.FieldDataType;
import org.elasticsearch.index.field.data.NumericFieldData;
import org.elasticsearch.index.mapper.FieldMapper;
import org.elasticsearch.index.mapper.MapperService;
import org.elasticsearch.script.SearchScript;
import org.elasticsearch.search.facet.AbstractFacetCollector;
import org.elasticsearch.search.facet.Facet;
import org.elasticsearch.search.facet.termsstats.TermsStatsFacet;
import org.elasticsearch.search.internal.SearchContext;
import java.io.IOException;
import java.util.ArrayDeque;
import java.util.Deque;
import java.util.Map;
import java.util.TreeSet;
public class TermsStatsDoubleFacetCollector extends AbstractFacetCollector {
private final TermsStatsFacet.ComparatorType comparatorType;
private final FieldDataCache fieldDataCache;
private final String keyFieldName;
private final String valueFieldName;
private final int size;
private final int numberOfShards;
private final FieldDataType keyFieldDataType;
private NumericFieldData keyFieldData;
private final FieldDataType valueFieldDataType;
private NumericFieldData valueFieldData;
private final SearchScript script;
private int missing = 0;
private final TDoubleObjectHashMap<InternalTermsStatsDoubleFacet.DoubleEntry> entries;
public TermsStatsDoubleFacetCollector(String facetName, String keyFieldName, String valueFieldName, int size, TermsStatsFacet.ComparatorType comparatorType,
SearchContext context, String scriptLang, String script, Map<String, Object> params) {
super(facetName);
this.fieldDataCache = context.fieldDataCache();
this.size = size;
this.comparatorType = comparatorType;
this.numberOfShards = context.numberOfShards();
MapperService.SmartNameFieldMappers smartMappers = context.mapperService().smartName(keyFieldName);
if (smartMappers == null || !smartMappers.hasMapper()) {
this.keyFieldName = keyFieldName;
this.keyFieldDataType = FieldDataType.DefaultTypes.STRING;
} else {
// add type filter if there is exact doc mapper associated with it
if (smartMappers.hasDocMapper()) {
setFilter(context.filterCache().cache(smartMappers.docMapper().typeFilter()));
}
this.keyFieldName = smartMappers.mapper().names().indexName();
this.keyFieldDataType = smartMappers.mapper().fieldDataType();
}
if (script == null) {
FieldMapper fieldMapper = context.mapperService().smartNameFieldMapper(valueFieldName);
if (fieldMapper == null) {
throw new ElasticSearchIllegalArgumentException("failed to find mappings for [" + valueFieldName + "]");
}
this.valueFieldName = fieldMapper.names().indexName();
this.valueFieldDataType = fieldMapper.fieldDataType();
this.script = null;
} else {
this.valueFieldName = null;
this.valueFieldDataType = null;
this.script = context.scriptService().search(context.lookup(), scriptLang, script, params);
}
this.entries = popFacets();
}
@Override public void setScorer(Scorer scorer) throws IOException {
if (script != null) {
script.setScorer(scorer);
}
}
@Override protected void doSetNextReader(IndexReader reader, int docBase) throws IOException {
keyFieldData = (NumericFieldData) fieldDataCache.cache(keyFieldDataType, reader, keyFieldName);
if (valueFieldName != null) {
valueFieldData = (NumericFieldData) fieldDataCache.cache(valueFieldDataType, reader, valueFieldName);
}
if (script != null) {
script.setNextReader(reader);
}
}
@Override protected void doCollect(int doc) throws IOException {
if (!keyFieldData.hasValue(doc)) {
missing++;
return;
}
double key = keyFieldData.doubleValue(doc);
InternalTermsStatsDoubleFacet.DoubleEntry DoubleEntry = entries.get(key);
if (DoubleEntry == null) {
DoubleEntry = new InternalTermsStatsDoubleFacet.DoubleEntry(key, 1, 0);
entries.put(key, DoubleEntry);
} else {
DoubleEntry.count++;
}
if (script == null) {
if (valueFieldData.multiValued()) {
for (double value : valueFieldData.doubleValues(doc)) {
DoubleEntry.total += value;
}
} else {
double value = valueFieldData.doubleValue(doc);
DoubleEntry.total += value;
}
} else {
script.setNextDocId(doc);
double value = script.runAsDouble();
DoubleEntry.total += value;
}
}
@Override public Facet facet() {
if (entries.isEmpty()) {
return new InternalTermsStatsDoubleFacet(facetName, comparatorType, size, ImmutableList.<InternalTermsStatsDoubleFacet.DoubleEntry>of(), missing);
}
if (size == 0) { // all terms
// all terms, just return the collection, we will sort it on the way back
return new InternalTermsStatsDoubleFacet(facetName, comparatorType, 0 /* indicates all terms*/, entries.valueCollection(), missing);
}
// we need to fetch facets of "size * numberOfShards" because of problems in how they are distributed across shards
TreeSet<InternalTermsStatsDoubleFacet.DoubleEntry> ordered = new BoundedTreeSet<InternalTermsStatsDoubleFacet.DoubleEntry>(comparatorType.comparator(), size * numberOfShards);
ordered.addAll(entries.valueCollection());
// that's fine to push here, this thread will be released AFTER the entries have either been serialized
// or processed
pushFacets(entries);
return new InternalTermsStatsDoubleFacet(facetName, comparatorType, size, ordered, missing);
}
static TDoubleObjectHashMap<InternalTermsStatsDoubleFacet.DoubleEntry> popFacets() {
Deque<TDoubleObjectHashMap<InternalTermsStatsDoubleFacet.DoubleEntry>> deque = cache.get().get();
if (deque.isEmpty()) {
deque.add(new TDoubleObjectHashMap<InternalTermsStatsDoubleFacet.DoubleEntry>());
}
TDoubleObjectHashMap<InternalTermsStatsDoubleFacet.DoubleEntry> facets = deque.pollFirst();
facets.clear();
return facets;
}
static void pushFacets(TDoubleObjectHashMap<InternalTermsStatsDoubleFacet.DoubleEntry> facets) {
facets.clear();
Deque<TDoubleObjectHashMap<InternalTermsStatsDoubleFacet.DoubleEntry>> deque = cache.get().get();
if (deque != null) {
deque.add(facets);
}
}
static ThreadLocal<ThreadLocals.CleanableValue<Deque<TDoubleObjectHashMap<InternalTermsStatsDoubleFacet.DoubleEntry>>>> cache = new ThreadLocal<ThreadLocals.CleanableValue<Deque<TDoubleObjectHashMap<InternalTermsStatsDoubleFacet.DoubleEntry>>>>() {
@Override protected ThreadLocals.CleanableValue<Deque<TDoubleObjectHashMap<InternalTermsStatsDoubleFacet.DoubleEntry>>> initialValue() {
return new ThreadLocals.CleanableValue<Deque<TDoubleObjectHashMap<InternalTermsStatsDoubleFacet.DoubleEntry>>>(new ArrayDeque<TDoubleObjectHashMap<InternalTermsStatsDoubleFacet.DoubleEntry>>());
}
};
}

View File

@ -0,0 +1,285 @@
/*
* Licensed to Elastic Search and Shay Banon under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. Elastic Search 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.elasticsearch.search.facet.termsstats.longs;
import org.elasticsearch.common.collect.BoundedTreeSet;
import org.elasticsearch.common.collect.ImmutableList;
import org.elasticsearch.common.io.stream.StreamInput;
import org.elasticsearch.common.io.stream.StreamOutput;
import org.elasticsearch.common.thread.ThreadLocals;
import org.elasticsearch.common.trove.map.hash.THashMap;
import org.elasticsearch.common.xcontent.XContentBuilder;
import org.elasticsearch.common.xcontent.XContentBuilderString;
import org.elasticsearch.search.facet.Facet;
import org.elasticsearch.search.facet.termsstats.InternalTermsStatsFacet;
import java.io.IOException;
import java.util.*;
public class InternalTermsStatsLongFacet extends InternalTermsStatsFacet {
private static final String STREAM_TYPE = "lTS";
public static void registerStream() {
Streams.registerStream(STREAM, STREAM_TYPE);
}
static Stream STREAM = new Stream() {
@Override public Facet readFacet(String type, StreamInput in) throws IOException {
return readTermsStatsFacet(in);
}
};
@Override public String streamType() {
return STREAM_TYPE;
}
public InternalTermsStatsLongFacet() {
}
public static class LongEntry implements Entry {
long term;
int count;
double total;
public LongEntry(long term, int count, double total) {
this.term = term;
this.count = count;
this.total = total;
}
@Override public String term() {
return Long.toString(term);
}
@Override public String getTerm() {
return term();
}
@Override public Number termAsNumber() {
return term;
}
@Override public Number getTermAsNumber() {
return termAsNumber();
}
@Override public int count() {
return count;
}
@Override public int getCount() {
return count();
}
@Override public double total() {
return total;
}
@Override public double getTotal() {
return total();
}
@Override public double mean() {
return total / count;
}
@Override public double getMean() {
return mean();
}
@Override public int compareTo(Entry o) {
LongEntry other = (LongEntry) o;
int i = (term < other.term ? -1 : (term == other.term ? 0 : 1));
if (i == 0) {
i = count - o.count();
if (i == 0) {
i = System.identityHashCode(this) - System.identityHashCode(o);
}
}
return i;
}
}
private String name;
int requiredSize;
long missing;
Collection<LongEntry> entries = ImmutableList.of();
ComparatorType comparatorType;
public InternalTermsStatsLongFacet(String name, ComparatorType comparatorType, int requiredSize, Collection<LongEntry> entries, long missing) {
this.name = name;
this.comparatorType = comparatorType;
this.requiredSize = requiredSize;
this.entries = entries;
this.missing = missing;
}
@Override public String name() {
return this.name;
}
@Override public String getName() {
return this.name;
}
@Override public String type() {
return TYPE;
}
@Override public String getType() {
return type();
}
@Override public List<LongEntry> entries() {
if (!(entries instanceof List)) {
entries = ImmutableList.copyOf(entries);
}
return (List<LongEntry>) entries;
}
@Override public List<LongEntry> getEntries() {
return entries();
}
@SuppressWarnings({"unchecked"}) @Override public Iterator<Entry> iterator() {
return (Iterator) entries.iterator();
}
@Override public long missingCount() {
return this.missing;
}
@Override public long getMissingCount() {
return missingCount();
}
private static ThreadLocal<ThreadLocals.CleanableValue<THashMap<String, LongEntry>>> aggregateCache = new ThreadLocal<ThreadLocals.CleanableValue<THashMap<String, LongEntry>>>() {
@Override protected ThreadLocals.CleanableValue<THashMap<String, LongEntry>> initialValue() {
return new ThreadLocals.CleanableValue<THashMap<String, LongEntry>>(new THashMap<String, LongEntry>());
}
};
@Override public Facet reduce(String name, List<Facet> facets) {
if (facets.size() == 1) {
if (requiredSize == 0) {
// we need to sort it here!
InternalTermsStatsLongFacet tsFacet = (InternalTermsStatsLongFacet) facets.get(0);
List<LongEntry> entries = tsFacet.entries();
Collections.sort(entries, comparatorType.comparator());
}
return facets.get(0);
}
int missing = 0;
THashMap<String, LongEntry> map = aggregateCache.get().get();
map.clear();
for (Facet facet : facets) {
InternalTermsStatsLongFacet tsFacet = (InternalTermsStatsLongFacet) facet;
missing += tsFacet.missing;
for (Entry entry : tsFacet) {
LongEntry longEntry = (LongEntry) entry;
LongEntry current = map.get(longEntry.term());
if (current != null) {
current.count += longEntry.count;
current.total += longEntry.total;
} else {
map.put(longEntry.term(), longEntry);
}
}
}
// sort
if (requiredSize == 0) { // all terms
LongEntry[] entries1 = map.values().toArray(new LongEntry[map.size()]);
Arrays.sort(entries1, comparatorType.comparator());
return new InternalTermsStatsLongFacet(name, comparatorType, requiredSize, Arrays.asList(entries1), missing);
} else {
TreeSet<LongEntry> ordered = new BoundedTreeSet<LongEntry>(comparatorType.comparator(), requiredSize);
ordered.addAll(map.values());
return new InternalTermsStatsLongFacet(name, comparatorType, requiredSize, ordered, missing);
}
}
static final class Fields {
static final XContentBuilderString _TYPE = new XContentBuilderString("_type");
static final XContentBuilderString MISSING = new XContentBuilderString("missing");
static final XContentBuilderString TERMS = new XContentBuilderString("terms");
static final XContentBuilderString TERM = new XContentBuilderString("term");
static final XContentBuilderString COUNT = new XContentBuilderString("count");
static final XContentBuilderString TOTAL = new XContentBuilderString("total");
static final XContentBuilderString MEAN = new XContentBuilderString("mean");
}
@Override public XContentBuilder toXContent(XContentBuilder builder, Params params) throws IOException {
builder.startObject(name);
builder.field(Fields._TYPE, InternalTermsStatsFacet.TYPE);
builder.field(Fields.MISSING, missing);
builder.startArray(Fields.TERMS);
for (Entry entry : entries) {
builder.startObject();
builder.field(Fields.TERM, ((LongEntry) entry).term);
builder.field(Fields.COUNT, entry.count());
builder.field(Fields.TOTAL, entry.total());
builder.field(Fields.MEAN, entry.mean());
builder.endObject();
}
builder.endArray();
builder.endObject();
return builder;
}
public static InternalTermsStatsLongFacet readTermsStatsFacet(StreamInput in) throws IOException {
InternalTermsStatsLongFacet facet = new InternalTermsStatsLongFacet();
facet.readFrom(in);
return facet;
}
@Override public void readFrom(StreamInput in) throws IOException {
name = in.readUTF();
comparatorType = ComparatorType.fromId(in.readByte());
requiredSize = in.readVInt();
missing = in.readVLong();
int size = in.readVInt();
entries = new ArrayList<LongEntry>(size);
for (int i = 0; i < size; i++) {
entries.add(new LongEntry(in.readLong(), in.readVInt(), in.readDouble()));
}
}
@Override public void writeTo(StreamOutput out) throws IOException {
out.writeUTF(name);
out.writeByte(comparatorType.id());
out.writeVInt(requiredSize);
out.writeVLong(missing);
out.writeVInt(entries.size());
for (Entry entry : entries) {
out.writeLong(((LongEntry) entry).term);
out.writeVInt(entry.count());
out.writeDouble(entry.total());
}
}
}

View File

@ -0,0 +1,200 @@
/*
* Licensed to Elastic Search and Shay Banon under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. Elastic Search 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.elasticsearch.search.facet.termsstats.longs;
import org.apache.lucene.index.IndexReader;
import org.apache.lucene.search.Scorer;
import org.elasticsearch.ElasticSearchIllegalArgumentException;
import org.elasticsearch.common.collect.BoundedTreeSet;
import org.elasticsearch.common.collect.ImmutableList;
import org.elasticsearch.common.thread.ThreadLocals;
import org.elasticsearch.common.trove.map.hash.TLongObjectHashMap;
import org.elasticsearch.index.cache.field.data.FieldDataCache;
import org.elasticsearch.index.field.data.FieldDataType;
import org.elasticsearch.index.field.data.NumericFieldData;
import org.elasticsearch.index.mapper.FieldMapper;
import org.elasticsearch.index.mapper.MapperService;
import org.elasticsearch.script.SearchScript;
import org.elasticsearch.search.facet.AbstractFacetCollector;
import org.elasticsearch.search.facet.Facet;
import org.elasticsearch.search.facet.termsstats.TermsStatsFacet;
import org.elasticsearch.search.internal.SearchContext;
import java.io.IOException;
import java.util.ArrayDeque;
import java.util.Deque;
import java.util.Map;
import java.util.TreeSet;
public class TermsStatsLongFacetCollector extends AbstractFacetCollector {
private final TermsStatsFacet.ComparatorType comparatorType;
private final FieldDataCache fieldDataCache;
private final String keyFieldName;
private final String valueFieldName;
private final int size;
private final int numberOfShards;
private final FieldDataType keyFieldDataType;
private NumericFieldData keyFieldData;
private final FieldDataType valueFieldDataType;
private NumericFieldData valueFieldData;
private final SearchScript script;
private int missing = 0;
private final TLongObjectHashMap<InternalTermsStatsLongFacet.LongEntry> entries;
public TermsStatsLongFacetCollector(String facetName, String keyFieldName, String valueFieldName, int size, TermsStatsFacet.ComparatorType comparatorType,
SearchContext context, String scriptLang, String script, Map<String, Object> params) {
super(facetName);
this.fieldDataCache = context.fieldDataCache();
this.size = size;
this.comparatorType = comparatorType;
this.numberOfShards = context.numberOfShards();
MapperService.SmartNameFieldMappers smartMappers = context.mapperService().smartName(keyFieldName);
if (smartMappers == null || !smartMappers.hasMapper()) {
this.keyFieldName = keyFieldName;
this.keyFieldDataType = FieldDataType.DefaultTypes.STRING;
} else {
// add type filter if there is exact doc mapper associated with it
if (smartMappers.hasDocMapper()) {
setFilter(context.filterCache().cache(smartMappers.docMapper().typeFilter()));
}
this.keyFieldName = smartMappers.mapper().names().indexName();
this.keyFieldDataType = smartMappers.mapper().fieldDataType();
}
if (script == null) {
FieldMapper fieldMapper = context.mapperService().smartNameFieldMapper(valueFieldName);
if (fieldMapper == null) {
throw new ElasticSearchIllegalArgumentException("failed to find mappings for [" + valueFieldName + "]");
}
this.valueFieldName = fieldMapper.names().indexName();
this.valueFieldDataType = fieldMapper.fieldDataType();
this.script = null;
} else {
this.valueFieldName = null;
this.valueFieldDataType = null;
this.script = context.scriptService().search(context.lookup(), scriptLang, script, params);
}
this.entries = popFacets();
}
@Override public void setScorer(Scorer scorer) throws IOException {
if (script != null) {
script.setScorer(scorer);
}
}
@Override protected void doSetNextReader(IndexReader reader, int docBase) throws IOException {
keyFieldData = (NumericFieldData) fieldDataCache.cache(keyFieldDataType, reader, keyFieldName);
if (valueFieldName != null) {
valueFieldData = (NumericFieldData) fieldDataCache.cache(valueFieldDataType, reader, valueFieldName);
}
if (script != null) {
script.setNextReader(reader);
}
}
@Override protected void doCollect(int doc) throws IOException {
if (!keyFieldData.hasValue(doc)) {
missing++;
return;
}
long key = keyFieldData.longValue(doc);
InternalTermsStatsLongFacet.LongEntry LongEntry = entries.get(key);
if (LongEntry == null) {
LongEntry = new InternalTermsStatsLongFacet.LongEntry(key, 1, 0);
entries.put(key, LongEntry);
} else {
LongEntry.count++;
}
if (script == null) {
if (valueFieldData.multiValued()) {
for (double value : valueFieldData.doubleValues(doc)) {
LongEntry.total += value;
}
} else {
double value = valueFieldData.doubleValue(doc);
LongEntry.total += value;
}
} else {
script.setNextDocId(doc);
double value = script.runAsDouble();
LongEntry.total += value;
}
}
@Override public Facet facet() {
if (entries.isEmpty()) {
return new InternalTermsStatsLongFacet(facetName, comparatorType, size, ImmutableList.<InternalTermsStatsLongFacet.LongEntry>of(), missing);
}
if (size == 0) { // all terms
// all terms, just return the collection, we will sort it on the way back
return new InternalTermsStatsLongFacet(facetName, comparatorType, 0 /* indicates all terms*/, entries.valueCollection(), missing);
}
// we need to fetch facets of "size * numberOfShards" because of problems in how they are distributed across shards
TreeSet<InternalTermsStatsLongFacet.LongEntry> ordered = new BoundedTreeSet<InternalTermsStatsLongFacet.LongEntry>(comparatorType.comparator(), size * numberOfShards);
ordered.addAll(entries.valueCollection());
// that's fine to push here, this thread will be released AFTER the entries have either been serialized
// or processed
pushFacets(entries);
return new InternalTermsStatsLongFacet(facetName, comparatorType, size, ordered, missing);
}
static TLongObjectHashMap<InternalTermsStatsLongFacet.LongEntry> popFacets() {
Deque<TLongObjectHashMap<InternalTermsStatsLongFacet.LongEntry>> deque = cache.get().get();
if (deque.isEmpty()) {
deque.add(new TLongObjectHashMap<InternalTermsStatsLongFacet.LongEntry>());
}
TLongObjectHashMap<InternalTermsStatsLongFacet.LongEntry> facets = deque.pollFirst();
facets.clear();
return facets;
}
static void pushFacets(TLongObjectHashMap<InternalTermsStatsLongFacet.LongEntry> facets) {
facets.clear();
Deque<TLongObjectHashMap<InternalTermsStatsLongFacet.LongEntry>> deque = cache.get().get();
if (deque != null) {
deque.add(facets);
}
}
static ThreadLocal<ThreadLocals.CleanableValue<Deque<TLongObjectHashMap<InternalTermsStatsLongFacet.LongEntry>>>> cache = new ThreadLocal<ThreadLocals.CleanableValue<Deque<TLongObjectHashMap<InternalTermsStatsLongFacet.LongEntry>>>>() {
@Override protected ThreadLocals.CleanableValue<Deque<TLongObjectHashMap<InternalTermsStatsLongFacet.LongEntry>>> initialValue() {
return new ThreadLocals.CleanableValue<Deque<TLongObjectHashMap<InternalTermsStatsLongFacet.LongEntry>>>(new ArrayDeque<TLongObjectHashMap<InternalTermsStatsLongFacet.LongEntry>>());
}
};
}

View File

@ -0,0 +1,286 @@
/*
* Licensed to Elastic Search and Shay Banon under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. Elastic Search 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.elasticsearch.search.facet.termsstats.strings;
import org.elasticsearch.common.collect.BoundedTreeSet;
import org.elasticsearch.common.collect.ImmutableList;
import org.elasticsearch.common.io.stream.StreamInput;
import org.elasticsearch.common.io.stream.StreamOutput;
import org.elasticsearch.common.thread.ThreadLocals;
import org.elasticsearch.common.trove.map.hash.THashMap;
import org.elasticsearch.common.xcontent.XContentBuilder;
import org.elasticsearch.common.xcontent.XContentBuilderString;
import org.elasticsearch.search.facet.Facet;
import org.elasticsearch.search.facet.termsstats.InternalTermsStatsFacet;
import java.io.IOException;
import java.util.*;
public class InternalTermsStatsStringFacet extends InternalTermsStatsFacet {
private static final String STREAM_TYPE = "tTS";
public static void registerStream() {
Streams.registerStream(STREAM, STREAM_TYPE);
}
static Stream STREAM = new Stream() {
@Override public Facet readFacet(String type, StreamInput in) throws IOException {
return readTermsStatsFacet(in);
}
};
@Override public String streamType() {
return STREAM_TYPE;
}
public InternalTermsStatsStringFacet() {
}
public static class StringEntry implements Entry {
String term;
int count;
double total;
public StringEntry(String term, int count, double total) {
this.term = term;
this.count = count;
this.total = total;
}
@Override public String term() {
return term;
}
@Override public String getTerm() {
return term();
}
@Override public Number termAsNumber() {
return Double.parseDouble(term);
}
@Override public Number getTermAsNumber() {
return termAsNumber();
}
@Override public int count() {
return count;
}
@Override public int getCount() {
return count();
}
@Override public double total() {
return total;
}
@Override public double getTotal() {
return total();
}
@Override public double mean() {
return total / count;
}
@Override public double getMean() {
return mean();
}
@Override public int compareTo(Entry o) {
int i = term.compareTo(o.term());
if (i == 0) {
i = count - o.count();
if (i == 0) {
i = System.identityHashCode(this) - System.identityHashCode(o);
}
}
return i;
}
}
private String name;
int requiredSize;
long missing;
Collection<StringEntry> entries = ImmutableList.of();
ComparatorType comparatorType;
public InternalTermsStatsStringFacet(String name, ComparatorType comparatorType, int requiredSize, Collection<StringEntry> entries, long missing) {
this.name = name;
this.comparatorType = comparatorType;
this.requiredSize = requiredSize;
this.entries = entries;
this.missing = missing;
}
@Override public String name() {
return this.name;
}
@Override public String getName() {
return this.name;
}
@Override public String type() {
return TYPE;
}
@Override public String getType() {
return type();
}
@Override public List<StringEntry> entries() {
if (!(entries instanceof List)) {
entries = ImmutableList.copyOf(entries);
}
return (List<StringEntry>) entries;
}
@Override public List<StringEntry> getEntries() {
return entries();
}
@SuppressWarnings({"unchecked"}) @Override public Iterator<Entry> iterator() {
return (Iterator) entries.iterator();
}
@Override public long missingCount() {
return this.missing;
}
@Override public long getMissingCount() {
return missingCount();
}
private static ThreadLocal<ThreadLocals.CleanableValue<THashMap<String, StringEntry>>> aggregateCache = new ThreadLocal<ThreadLocals.CleanableValue<THashMap<String, StringEntry>>>() {
@Override protected ThreadLocals.CleanableValue<THashMap<String, StringEntry>> initialValue() {
return new ThreadLocals.CleanableValue<THashMap<String, StringEntry>>(new THashMap<String, StringEntry>());
}
};
@Override public Facet reduce(String name, List<Facet> facets) {
if (facets.size() == 1) {
if (requiredSize == 0) {
// we need to sort it here!
InternalTermsStatsStringFacet tsFacet = (InternalTermsStatsStringFacet) facets.get(0);
List<StringEntry> entries = tsFacet.entries();
if (!entries.isEmpty()) {
Collections.sort(entries, comparatorType.comparator());
}
}
return facets.get(0);
}
int missing = 0;
THashMap<String, StringEntry> map = aggregateCache.get().get();
map.clear();
for (Facet facet : facets) {
InternalTermsStatsStringFacet tsFacet = (InternalTermsStatsStringFacet) facet;
missing += tsFacet.missing;
for (Entry entry : tsFacet) {
StringEntry stringEntry = (StringEntry) entry;
StringEntry current = map.get(stringEntry.term());
if (current != null) {
current.count += stringEntry.count;
current.total += stringEntry.total;
} else {
map.put(stringEntry.term(), stringEntry);
}
}
}
// sort
if (requiredSize == 0) { // all terms
StringEntry[] entries1 = map.values().toArray(new StringEntry[map.size()]);
Arrays.sort(entries1, comparatorType.comparator());
return new InternalTermsStatsStringFacet(name, comparatorType, requiredSize, Arrays.asList(entries1), missing);
} else {
TreeSet<StringEntry> ordered = new BoundedTreeSet<StringEntry>(comparatorType.comparator(), requiredSize);
ordered.addAll(map.values());
return new InternalTermsStatsStringFacet(name, comparatorType, requiredSize, ordered, missing);
}
}
static final class Fields {
static final XContentBuilderString _TYPE = new XContentBuilderString("_type");
static final XContentBuilderString MISSING = new XContentBuilderString("missing");
static final XContentBuilderString TERMS = new XContentBuilderString("terms");
static final XContentBuilderString TERM = new XContentBuilderString("term");
static final XContentBuilderString COUNT = new XContentBuilderString("count");
static final XContentBuilderString TOTAL = new XContentBuilderString("total");
static final XContentBuilderString MEAN = new XContentBuilderString("mean");
}
@Override public XContentBuilder toXContent(XContentBuilder builder, Params params) throws IOException {
builder.startObject(name);
builder.field(Fields._TYPE, InternalTermsStatsFacet.TYPE);
builder.field(Fields.MISSING, missing);
builder.startArray(Fields.TERMS);
for (Entry entry : entries) {
builder.startObject();
builder.field(Fields.TERM, entry.term());
builder.field(Fields.COUNT, entry.count());
builder.field(Fields.TOTAL, entry.total());
builder.field(Fields.MEAN, entry.mean());
builder.endObject();
}
builder.endArray();
builder.endObject();
return builder;
}
public static InternalTermsStatsStringFacet readTermsStatsFacet(StreamInput in) throws IOException {
InternalTermsStatsStringFacet facet = new InternalTermsStatsStringFacet();
facet.readFrom(in);
return facet;
}
@Override public void readFrom(StreamInput in) throws IOException {
name = in.readUTF();
comparatorType = ComparatorType.fromId(in.readByte());
requiredSize = in.readVInt();
missing = in.readVLong();
int size = in.readVInt();
entries = new ArrayList<StringEntry>(size);
for (int i = 0; i < size; i++) {
entries.add(new StringEntry(in.readUTF(), in.readVInt(), in.readDouble()));
}
}
@Override public void writeTo(StreamOutput out) throws IOException {
out.writeUTF(name);
out.writeByte(comparatorType.id());
out.writeVInt(requiredSize);
out.writeVLong(missing);
out.writeVInt(entries.size());
for (Entry entry : entries) {
out.writeUTF(entry.term());
out.writeVInt(entry.count());
out.writeDouble(entry.total());
}
}
}

View File

@ -0,0 +1,201 @@
/*
* Licensed to Elastic Search and Shay Banon under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. Elastic Search 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.elasticsearch.search.facet.termsstats.strings;
import org.apache.lucene.index.IndexReader;
import org.apache.lucene.search.Scorer;
import org.elasticsearch.ElasticSearchIllegalArgumentException;
import org.elasticsearch.common.collect.BoundedTreeSet;
import org.elasticsearch.common.collect.ImmutableList;
import org.elasticsearch.common.thread.ThreadLocals;
import org.elasticsearch.common.trove.map.hash.THashMap;
import org.elasticsearch.index.cache.field.data.FieldDataCache;
import org.elasticsearch.index.field.data.FieldData;
import org.elasticsearch.index.field.data.FieldDataType;
import org.elasticsearch.index.field.data.NumericFieldData;
import org.elasticsearch.index.mapper.FieldMapper;
import org.elasticsearch.index.mapper.MapperService;
import org.elasticsearch.script.SearchScript;
import org.elasticsearch.search.facet.AbstractFacetCollector;
import org.elasticsearch.search.facet.Facet;
import org.elasticsearch.search.facet.termsstats.TermsStatsFacet;
import org.elasticsearch.search.internal.SearchContext;
import java.io.IOException;
import java.util.ArrayDeque;
import java.util.Deque;
import java.util.Map;
import java.util.TreeSet;
public class TermsStatsStringFacetCollector extends AbstractFacetCollector {
private final TermsStatsFacet.ComparatorType comparatorType;
private final FieldDataCache fieldDataCache;
private final String keyFieldName;
private final String valueFieldName;
private final int size;
private final int numberOfShards;
private final FieldDataType keyFieldDataType;
private FieldData keyFieldData;
private final FieldDataType valueFieldDataType;
private NumericFieldData valueFieldData;
private final SearchScript script;
private int missing = 0;
private final THashMap<String, InternalTermsStatsStringFacet.StringEntry> entries;
public TermsStatsStringFacetCollector(String facetName, String keyFieldName, String valueFieldName, int size, TermsStatsFacet.ComparatorType comparatorType,
SearchContext context, String scriptLang, String script, Map<String, Object> params) {
super(facetName);
this.fieldDataCache = context.fieldDataCache();
this.size = size;
this.comparatorType = comparatorType;
this.numberOfShards = context.numberOfShards();
MapperService.SmartNameFieldMappers smartMappers = context.mapperService().smartName(keyFieldName);
if (smartMappers == null || !smartMappers.hasMapper()) {
this.keyFieldName = keyFieldName;
this.keyFieldDataType = FieldDataType.DefaultTypes.STRING;
} else {
// add type filter if there is exact doc mapper associated with it
if (smartMappers.hasDocMapper()) {
setFilter(context.filterCache().cache(smartMappers.docMapper().typeFilter()));
}
this.keyFieldName = smartMappers.mapper().names().indexName();
this.keyFieldDataType = smartMappers.mapper().fieldDataType();
}
if (script == null) {
FieldMapper fieldMapper = context.mapperService().smartNameFieldMapper(valueFieldName);
if (fieldMapper == null) {
throw new ElasticSearchIllegalArgumentException("failed to find mappings for [" + valueFieldName + "]");
}
this.valueFieldName = fieldMapper.names().indexName();
this.valueFieldDataType = fieldMapper.fieldDataType();
this.script = null;
} else {
this.valueFieldName = null;
this.valueFieldDataType = null;
this.script = context.scriptService().search(context.lookup(), scriptLang, script, params);
}
this.entries = popFacets();
}
@Override public void setScorer(Scorer scorer) throws IOException {
if (script != null) {
script.setScorer(scorer);
}
}
@Override protected void doSetNextReader(IndexReader reader, int docBase) throws IOException {
keyFieldData = fieldDataCache.cache(keyFieldDataType, reader, keyFieldName);
if (valueFieldName != null) {
valueFieldData = (NumericFieldData) fieldDataCache.cache(valueFieldDataType, reader, valueFieldName);
}
if (script != null) {
script.setNextReader(reader);
}
}
@Override protected void doCollect(int doc) throws IOException {
if (!keyFieldData.hasValue(doc)) {
missing++;
return;
}
String key = keyFieldData.stringValue(doc);
InternalTermsStatsStringFacet.StringEntry stringEntry = entries.get(key);
if (stringEntry == null) {
stringEntry = new InternalTermsStatsStringFacet.StringEntry(key, 1, 0);
entries.put(key, stringEntry);
} else {
stringEntry.count++;
}
if (script == null) {
if (valueFieldData.multiValued()) {
for (double value : valueFieldData.doubleValues(doc)) {
stringEntry.total += value;
}
} else {
double value = valueFieldData.doubleValue(doc);
stringEntry.total += value;
}
} else {
script.setNextDocId(doc);
double value = script.runAsDouble();
stringEntry.total += value;
}
}
@Override public Facet facet() {
if (entries.isEmpty()) {
return new InternalTermsStatsStringFacet(facetName, comparatorType, size, ImmutableList.<InternalTermsStatsStringFacet.StringEntry>of(), missing);
}
if (size == 0) { // all terms
// all terms, just return the collection, we will sort it on the way back
return new InternalTermsStatsStringFacet(facetName, comparatorType, 0 /* indicates all terms*/, entries.values(), missing);
}
// we need to fetch facets of "size * numberOfShards" because of problems in how they are distributed across shards
TreeSet<InternalTermsStatsStringFacet.StringEntry> ordered = new BoundedTreeSet<InternalTermsStatsStringFacet.StringEntry>(comparatorType.comparator(), size * numberOfShards);
ordered.addAll(entries.values());
// that's fine to push here, this thread will be released AFTER the entries have either been serialized
// or processed
pushFacets(entries);
return new InternalTermsStatsStringFacet(facetName, comparatorType, size, ordered, missing);
}
static THashMap<String, InternalTermsStatsStringFacet.StringEntry> popFacets() {
Deque<THashMap<String, InternalTermsStatsStringFacet.StringEntry>> deque = cache.get().get();
if (deque.isEmpty()) {
deque.add(new THashMap<String, InternalTermsStatsStringFacet.StringEntry>());
}
THashMap<String, InternalTermsStatsStringFacet.StringEntry> facets = deque.pollFirst();
facets.clear();
return facets;
}
static void pushFacets(THashMap<String, InternalTermsStatsStringFacet.StringEntry> facets) {
facets.clear();
Deque<THashMap<String, InternalTermsStatsStringFacet.StringEntry>> deque = cache.get().get();
if (deque != null) {
deque.add(facets);
}
}
static ThreadLocal<ThreadLocals.CleanableValue<Deque<THashMap<String, InternalTermsStatsStringFacet.StringEntry>>>> cache = new ThreadLocal<ThreadLocals.CleanableValue<Deque<THashMap<String, InternalTermsStatsStringFacet.StringEntry>>>>() {
@Override protected ThreadLocals.CleanableValue<Deque<THashMap<String, InternalTermsStatsStringFacet.StringEntry>>> initialValue() {
return new ThreadLocals.CleanableValue<Deque<THashMap<String, InternalTermsStatsStringFacet.StringEntry>>>(new ArrayDeque<THashMap<String, InternalTermsStatsStringFacet.StringEntry>>());
}
};
}

View File

@ -40,6 +40,7 @@ import org.elasticsearch.search.facet.terms.doubles.InternalDoubleTermsFacet;
import org.elasticsearch.search.facet.terms.ints.InternalIntTermsFacet;
import org.elasticsearch.search.facet.terms.longs.InternalLongTermsFacet;
import org.elasticsearch.search.facet.terms.shorts.InternalShortTermsFacet;
import org.elasticsearch.search.facet.termsstats.TermsStatsFacet;
import org.elasticsearch.test.integration.AbstractNodesTests;
import org.testng.annotations.AfterClass;
import org.testng.annotations.BeforeClass;
@ -62,7 +63,7 @@ public class SimpleFacetsTests extends AbstractNodesTests {
private Client client;
@BeforeClass public void createNodes() throws Exception {
Settings settings = ImmutableSettings.settingsBuilder().put("number_of_shards", 3).put("number_of_replicas", 0).build();
Settings settings = ImmutableSettings.settingsBuilder().put("index.number_of_shards", 3).put("index.number_of_replicas", 0).build();
startNode("server1", settings);
startNode("server2", settings);
client = getClient();
@ -271,6 +272,13 @@ public class SimpleFacetsTests extends AbstractNodesTests {
assertThat(facet.entries().get(0).count(), equalTo(2));
assertThat(facet.entries().get(1).term(), equalTo("test2"));
assertThat(facet.entries().get(1).count(), equalTo(1));
try {
client.admin().indices().prepareDelete("test1").execute().actionGet();
client.admin().indices().prepareDelete("test2").execute().actionGet();
} catch (Exception e) {
// ignore
}
}
@Test public void testFilterFacets() throws Exception {
@ -1266,6 +1274,239 @@ public class SimpleFacetsTests extends AbstractNodesTests {
assertThat(facet.entries().get(1).total(), equalTo(10d));
}
@Test public void testTermsStatsFacets() throws Exception {
try {
client.admin().indices().prepareDelete("test").execute().actionGet();
} catch (Exception e) {
// ignore
}
client.admin().indices().prepareCreate("test").execute().actionGet();
client.admin().cluster().prepareHealth().setWaitForGreenStatus().execute().actionGet();
client.prepareIndex("test", "type1").setSource(jsonBuilder().startObject()
.field("field", "xxx")
.field("num", 100.0)
.startArray("multi_num").value(1.0).value(2.0f).endArray()
.endObject()).execute().actionGet();
client.prepareIndex("test", "type1").setSource(jsonBuilder().startObject()
.field("field", "xxx")
.field("num", 200.0)
.startArray("multi_num").value(2.0).value(3.0f).endArray()
.endObject()).execute().actionGet();
client.prepareIndex("test", "type1").setSource(jsonBuilder().startObject()
.field("field", "yyy")
.field("num", 500.0)
.startArray("multi_num").value(5.0).value(6.0f).endArray()
.endObject()).execute().actionGet();
client.admin().indices().prepareFlush().setRefresh(true).execute().actionGet();
SearchResponse searchResponse = client.prepareSearch()
.setQuery(matchAllQuery())
.addFacet(termsStats("stats1").keyField("field").valueField("num"))
.addFacet(termsStats("stats2").keyField("field").valueField("multi_num"))
.addFacet(termsStats("stats3").keyField("field").valueField("num").order(TermsStatsFacet.ComparatorType.COUNT))
.addFacet(termsStats("stats4").keyField("field").valueField("multi_num").order(TermsStatsFacet.ComparatorType.COUNT))
.addFacet(termsStats("stats5").keyField("field").valueField("num").order(TermsStatsFacet.ComparatorType.TOTAL))
.addFacet(termsStats("stats6").keyField("field").valueField("multi_num").order(TermsStatsFacet.ComparatorType.TOTAL))
.addFacet(termsStats("stats7").keyField("field").valueField("num").allTerms())
.addFacet(termsStats("stats8").keyField("field").valueField("multi_num").allTerms())
.addFacet(termsStats("stats9").keyField("field").valueField("num").order(TermsStatsFacet.ComparatorType.COUNT).allTerms())
.addFacet(termsStats("stats10").keyField("field").valueField("multi_num").order(TermsStatsFacet.ComparatorType.COUNT).allTerms())
.addFacet(termsStats("stats11").keyField("field").valueField("num").order(TermsStatsFacet.ComparatorType.TOTAL).allTerms())
.addFacet(termsStats("stats12").keyField("field").valueField("multi_num").order(TermsStatsFacet.ComparatorType.TOTAL).allTerms())
.addFacet(termsStats("stats13").keyField("field").valueScript("doc['num'].value * 2"))
.execute().actionGet();
if (searchResponse.failedShards() > 0) {
logger.warn("Failed shards:");
for (ShardSearchFailure shardSearchFailure : searchResponse.shardFailures()) {
logger.warn("-> {}", shardSearchFailure);
}
}
assertThat(searchResponse.failedShards(), equalTo(0));
TermsStatsFacet facet = searchResponse.facets().facet("stats1");
assertThat(facet.entries().size(), equalTo(2));
assertThat(facet.entries().get(0).term(), equalTo("xxx"));
assertThat(facet.entries().get(0).count(), equalTo(2));
assertThat(facet.entries().get(0).total(), closeTo(300d, 0.00001d));
assertThat(facet.entries().get(1).term(), equalTo("yyy"));
assertThat(facet.entries().get(1).count(), equalTo(1));
assertThat(facet.entries().get(1).total(), closeTo(500d, 0.00001d));
facet = searchResponse.facets().facet("stats2");
assertThat(facet.entries().size(), equalTo(2));
assertThat(facet.entries().get(0).term(), equalTo("xxx"));
assertThat(facet.entries().get(0).count(), equalTo(2));
assertThat(facet.entries().get(0).total(), closeTo(8d, 0.00001d));
assertThat(facet.entries().get(1).term(), equalTo("yyy"));
assertThat(facet.entries().get(1).count(), equalTo(1));
assertThat(facet.entries().get(1).total(), closeTo(11d, 0.00001d));
facet = searchResponse.facets().facet("stats3");
assertThat(facet.entries().size(), equalTo(2));
assertThat(facet.entries().get(0).term(), equalTo("xxx"));
assertThat(facet.entries().get(0).count(), equalTo(2));
assertThat(facet.entries().get(0).total(), closeTo(300d, 0.00001d));
assertThat(facet.entries().get(1).term(), equalTo("yyy"));
assertThat(facet.entries().get(1).count(), equalTo(1));
assertThat(facet.entries().get(1).total(), closeTo(500d, 0.00001d));
facet = searchResponse.facets().facet("stats4");
assertThat(facet.entries().size(), equalTo(2));
assertThat(facet.entries().get(0).term(), equalTo("xxx"));
assertThat(facet.entries().get(0).count(), equalTo(2));
assertThat(facet.entries().get(0).total(), closeTo(8d, 0.00001d));
assertThat(facet.entries().get(1).term(), equalTo("yyy"));
assertThat(facet.entries().get(1).count(), equalTo(1));
assertThat(facet.entries().get(1).total(), closeTo(11d, 0.00001d));
facet = searchResponse.facets().facet("stats5");
assertThat(facet.entries().size(), equalTo(2));
assertThat(facet.entries().get(0).term(), equalTo("yyy"));
assertThat(facet.entries().get(0).count(), equalTo(1));
assertThat(facet.entries().get(0).total(), closeTo(500d, 0.00001d));
assertThat(facet.entries().get(1).term(), equalTo("xxx"));
assertThat(facet.entries().get(1).count(), equalTo(2));
assertThat(facet.entries().get(1).total(), closeTo(300d, 0.00001d));
facet = searchResponse.facets().facet("stats6");
assertThat(facet.entries().size(), equalTo(2));
assertThat(facet.entries().get(0).term(), equalTo("yyy"));
assertThat(facet.entries().get(0).count(), equalTo(1));
assertThat(facet.entries().get(0).total(), closeTo(11d, 0.00001d));
assertThat(facet.entries().get(1).term(), equalTo("xxx"));
assertThat(facet.entries().get(1).count(), equalTo(2));
assertThat(facet.entries().get(1).total(), closeTo(8d, 0.00001d));
facet = searchResponse.facets().facet("stats7");
assertThat(facet.entries().size(), equalTo(2));
assertThat(facet.entries().get(0).term(), equalTo("xxx"));
assertThat(facet.entries().get(0).count(), equalTo(2));
assertThat(facet.entries().get(0).total(), closeTo(300d, 0.00001d));
assertThat(facet.entries().get(1).term(), equalTo("yyy"));
assertThat(facet.entries().get(1).count(), equalTo(1));
assertThat(facet.entries().get(1).total(), closeTo(500d, 0.00001d));
facet = searchResponse.facets().facet("stats8");
assertThat(facet.entries().size(), equalTo(2));
assertThat(facet.entries().get(0).term(), equalTo("xxx"));
assertThat(facet.entries().get(0).count(), equalTo(2));
assertThat(facet.entries().get(0).total(), closeTo(8d, 0.00001d));
assertThat(facet.entries().get(1).term(), equalTo("yyy"));
assertThat(facet.entries().get(1).count(), equalTo(1));
assertThat(facet.entries().get(1).total(), closeTo(11d, 0.00001d));
facet = searchResponse.facets().facet("stats9");
assertThat(facet.entries().size(), equalTo(2));
assertThat(facet.entries().get(0).term(), equalTo("xxx"));
assertThat(facet.entries().get(0).count(), equalTo(2));
assertThat(facet.entries().get(0).total(), closeTo(300d, 0.00001d));
assertThat(facet.entries().get(1).term(), equalTo("yyy"));
assertThat(facet.entries().get(1).count(), equalTo(1));
assertThat(facet.entries().get(1).total(), closeTo(500d, 0.00001d));
facet = searchResponse.facets().facet("stats10");
assertThat(facet.entries().size(), equalTo(2));
assertThat(facet.entries().get(0).term(), equalTo("xxx"));
assertThat(facet.entries().get(0).count(), equalTo(2));
assertThat(facet.entries().get(0).total(), closeTo(8d, 0.00001d));
assertThat(facet.entries().get(1).term(), equalTo("yyy"));
assertThat(facet.entries().get(1).count(), equalTo(1));
assertThat(facet.entries().get(1).total(), closeTo(11d, 0.00001d));
facet = searchResponse.facets().facet("stats11");
assertThat(facet.entries().size(), equalTo(2));
assertThat(facet.entries().get(0).term(), equalTo("yyy"));
assertThat(facet.entries().get(0).count(), equalTo(1));
assertThat(facet.entries().get(0).total(), closeTo(500d, 0.00001d));
assertThat(facet.entries().get(1).term(), equalTo("xxx"));
assertThat(facet.entries().get(1).count(), equalTo(2));
assertThat(facet.entries().get(1).total(), closeTo(300d, 0.00001d));
facet = searchResponse.facets().facet("stats12");
assertThat(facet.entries().size(), equalTo(2));
assertThat(facet.entries().get(0).term(), equalTo("yyy"));
assertThat(facet.entries().get(0).count(), equalTo(1));
assertThat(facet.entries().get(0).total(), closeTo(11d, 0.00001d));
assertThat(facet.entries().get(1).term(), equalTo("xxx"));
assertThat(facet.entries().get(1).count(), equalTo(2));
assertThat(facet.entries().get(1).total(), closeTo(8d, 0.00001d));
facet = searchResponse.facets().facet("stats13");
assertThat(facet.entries().size(), equalTo(2));
assertThat(facet.entries().get(0).term(), equalTo("xxx"));
assertThat(facet.entries().get(0).count(), equalTo(2));
assertThat(facet.entries().get(0).total(), closeTo(600d, 0.00001d));
assertThat(facet.entries().get(1).term(), equalTo("yyy"));
assertThat(facet.entries().get(1).count(), equalTo(1));
assertThat(facet.entries().get(1).total(), closeTo(1000d, 0.00001d));
}
@Test public void testNumericTermsStatsFacets() throws Exception {
try {
client.admin().indices().prepareDelete("test").execute().actionGet();
} catch (Exception e) {
// ignore
}
client.admin().indices().prepareCreate("test").execute().actionGet();
client.admin().cluster().prepareHealth().setWaitForGreenStatus().execute().actionGet();
client.prepareIndex("test", "type1").setSource(jsonBuilder().startObject()
.field("lField", 100l)
.field("dField", 100.1d)
.field("num", 100.0)
.startArray("multi_num").value(1.0).value(2.0f).endArray()
.endObject()).execute().actionGet();
client.prepareIndex("test", "type1").setSource(jsonBuilder().startObject()
.field("lField", 100l)
.field("dField", 100.1d)
.field("num", 200.0)
.startArray("multi_num").value(2.0).value(3.0f).endArray()
.endObject()).execute().actionGet();
client.prepareIndex("test", "type1").setSource(jsonBuilder().startObject()
.field("lField", 200l)
.field("dField", 200.2d)
.field("num", 500.0)
.startArray("multi_num").value(5.0).value(6.0f).endArray()
.endObject()).execute().actionGet();
client.admin().indices().prepareFlush().setRefresh(true).execute().actionGet();
SearchResponse searchResponse = client.prepareSearch()
.setQuery(matchAllQuery())
.addFacet(termsStats("stats1").keyField("lField").valueField("num"))
.addFacet(termsStats("stats2").keyField("dField").valueField("num"))
.execute().actionGet();
if (searchResponse.failedShards() > 0) {
logger.warn("Failed shards:");
for (ShardSearchFailure shardSearchFailure : searchResponse.shardFailures()) {
logger.warn("-> {}", shardSearchFailure);
}
}
assertThat(searchResponse.failedShards(), equalTo(0));
TermsStatsFacet facet = searchResponse.facets().facet("stats1");
assertThat(facet.entries().size(), equalTo(2));
assertThat(facet.entries().get(0).term(), equalTo("100"));
assertThat(facet.entries().get(0).count(), equalTo(2));
assertThat(facet.entries().get(0).total(), closeTo(300d, 0.00001d));
assertThat(facet.entries().get(1).term(), equalTo("200"));
assertThat(facet.entries().get(1).count(), equalTo(1));
assertThat(facet.entries().get(1).total(), closeTo(500d, 0.00001d));
facet = searchResponse.facets().facet("stats2");
assertThat(facet.entries().size(), equalTo(2));
assertThat(facet.entries().get(0).term(), equalTo("100.1"));
assertThat(facet.entries().get(0).count(), equalTo(2));
assertThat(facet.entries().get(0).total(), closeTo(300d, 0.00001d));
assertThat(facet.entries().get(1).term(), equalTo("200.2"));
assertThat(facet.entries().get(1).count(), equalTo(1));
assertThat(facet.entries().get(1).total(), closeTo(500d, 0.00001d));
}
private long utcTimeInMillis(String time) {
return timeInMillis(time, DateTimeZone.UTC);
}