Browse Source

Add suggest stats

closes #4032
Kevin Wang 11 years ago
parent
commit
ceed22fe00
21 changed files with 496 additions and 6 deletions
  1. 1 0
      docs/reference/indices/stats.asciidoc
  2. 1 1
      rest-api-spec/api/indices.stats.json
  3. 1 1
      rest-api-spec/api/nodes.stats.json
  4. 36 1
      src/main/java/org/elasticsearch/action/admin/indices/stats/CommonStats.java
  5. 2 1
      src/main/java/org/elasticsearch/action/admin/indices/stats/CommonStatsFlags.java
  6. 9 0
      src/main/java/org/elasticsearch/action/admin/indices/stats/IndicesStatsRequest.java
  7. 5 0
      src/main/java/org/elasticsearch/action/admin/indices/stats/IndicesStatsRequestBuilder.java
  8. 3 0
      src/main/java/org/elasticsearch/action/admin/indices/stats/TransportIndicesStatsAction.java
  9. 5 0
      src/main/java/org/elasticsearch/action/suggest/TransportSuggestAction.java
  10. 2 0
      src/main/java/org/elasticsearch/index/service/InternalIndexService.java
  11. 6 0
      src/main/java/org/elasticsearch/index/shard/service/IndexShard.java
  12. 15 1
      src/main/java/org/elasticsearch/index/shard/service/InternalIndexShard.java
  13. 33 0
      src/main/java/org/elasticsearch/index/suggest/SuggestShardModule.java
  14. 68 0
      src/main/java/org/elasticsearch/index/suggest/stats/ShardSuggestService.java
  15. 124 0
      src/main/java/org/elasticsearch/index/suggest/stats/SuggestStats.java
  16. 6 0
      src/main/java/org/elasticsearch/indices/NodeIndicesStats.java
  17. 1 0
      src/main/java/org/elasticsearch/rest/action/admin/indices/stats/RestIndicesStatsAction.java
  18. 19 0
      src/main/java/org/elasticsearch/rest/action/cat/RestIndicesAction.java
  19. 8 0
      src/main/java/org/elasticsearch/rest/action/cat/RestNodesAction.java
  20. 145 0
      src/test/java/org/elasticsearch/index/suggest/stats/SuggestStatsTests.java
  21. 6 1
      src/test/java/org/elasticsearch/indices/stats/SimpleIndexStatsTests.java

+ 1 - 0
docs/reference/indices/stats.asciidoc

@@ -45,6 +45,7 @@ specified as well in the URI. Those stats can be any of:
 `flush`:: 		Flush statistics.
 `completion`:: 		Completion suggest statistics.
 `refresh`:: 	Refresh statistics.
+`suggest`:: 	Suggest statistics.
 
 Some statistics allow per field granularity which accepts a list comma-separated list of included fields. By default all fields are included:
 

+ 1 - 1
rest-api-spec/api/indices.stats.json

@@ -17,7 +17,7 @@
         },
         "metric" : {
           "type" : "list",
-          "options" : ["_all", "completion", "docs", "fielddata", "filter_cache", "flush", "get", "id_cache", "indexing", "merge", "percolate", "refresh", "search", "segments", "store", "warmer"],
+          "options" : ["_all", "completion", "docs", "fielddata", "filter_cache", "flush", "get", "id_cache", "indexing", "merge", "percolate", "refresh", "search", "segments", "store", "warmer", "suggest"],
           "description" : "Limit the information returned the specific metrics."
         }
       },

+ 1 - 1
rest-api-spec/api/nodes.stats.json

@@ -20,7 +20,7 @@
         },
         "index_metric" : {
           "type" : "list",
-          "options" : ["_all", "completion", "docs", "fielddata", "filter_cache", "flush", "get", "id_cache", "indexing", "merge", "percolate", "refresh", "search", "segments", "store", "warmer"],
+          "options" : ["_all", "completion", "docs", "fielddata", "filter_cache", "flush", "get", "id_cache", "indexing", "merge", "percolate", "refresh", "search", "segments", "store", "warmer", "suggest"],
           "description" : "Limit the information returned for `indices` metric to the specific index metrics. Isn't used if `indices` (or `all`) metric isn't specified."
         },
         "node_id": {

+ 36 - 1
src/main/java/org/elasticsearch/action/admin/indices/stats/CommonStats.java

@@ -19,6 +19,7 @@
 
 package org.elasticsearch.action.admin.indices.stats;
 
+import org.elasticsearch.Version;
 import org.elasticsearch.common.Nullable;
 import org.elasticsearch.common.io.stream.StreamInput;
 import org.elasticsearch.common.io.stream.StreamOutput;
@@ -39,6 +40,7 @@ import org.elasticsearch.index.search.stats.SearchStats;
 import org.elasticsearch.index.shard.DocsStats;
 import org.elasticsearch.index.shard.service.IndexShard;
 import org.elasticsearch.index.store.StoreStats;
+import org.elasticsearch.index.suggest.stats.SuggestStats;
 import org.elasticsearch.index.translog.TranslogStats;
 import org.elasticsearch.index.warmer.WarmerStats;
 import org.elasticsearch.search.suggest.completion.CompletionStats;
@@ -106,6 +108,9 @@ public class CommonStats implements Streamable, ToXContent {
                 case Translog:
                     translog = new TranslogStats();
                     break;
+                case Suggest:
+                    suggest = new SuggestStats();
+                    break;
                 default:
                     throw new IllegalStateException("Unknown Flag: " + flag);
             }
@@ -166,6 +171,9 @@ public class CommonStats implements Streamable, ToXContent {
                 case Translog:
                     translog = indexShard.translogStats();
                     break;
+                case Suggest:
+                    suggest = indexShard.suggestStats();
+                    break;
                 default:
                     throw new IllegalStateException("Unknown Flag: " + flag);
             }
@@ -220,6 +228,9 @@ public class CommonStats implements Streamable, ToXContent {
     @Nullable
     public TranslogStats translog;
 
+    @Nullable
+    public SuggestStats suggest;
+
     public void add(CommonStats stats) {
         if (docs == null) {
             if (stats.getDocs() != null) {
@@ -351,6 +362,14 @@ public class CommonStats implements Streamable, ToXContent {
         } else {
             translog.add(stats.getTranslog());
         }
+        if (suggest == null) {
+            if (stats.getSuggest() != null) {
+                suggest = new SuggestStats();
+                suggest.add(stats.getSuggest());
+            }
+        } else {
+            suggest.add(stats.getSuggest());
+        }
     }
 
     @Nullable
@@ -429,7 +448,14 @@ public class CommonStats implements Streamable, ToXContent {
     }
 
     @Nullable
-    public TranslogStats getTranslog() { return translog; }
+    public TranslogStats getTranslog() {
+        return translog;
+    }
+
+    @Nullable
+    public SuggestStats getSuggest() {
+        return suggest;
+    }
 
     public static CommonStats readCommonStats(StreamInput in) throws IOException {
         CommonStats stats = new CommonStats();
@@ -485,6 +511,9 @@ public class CommonStats implements Streamable, ToXContent {
             segments = SegmentsStats.readSegmentsStats(in);
         }
         translog = in.readOptionalStreamable(new TranslogStats());
+        if (in.getVersion().onOrAfter(Version.V_1_2_0)) {
+            suggest = in.readOptionalStreamable(new SuggestStats());
+        }
     }
 
     @Override
@@ -580,6 +609,9 @@ public class CommonStats implements Streamable, ToXContent {
             segments.writeTo(out);
         }
         out.writeOptionalStreamable(translog);
+        if (out.getVersion().onOrAfter(Version.V_1_2_0)) {
+            out.writeOptionalStreamable(suggest);
+        }
     }
 
     // note, requires a wrapping object
@@ -633,6 +665,9 @@ public class CommonStats implements Streamable, ToXContent {
         if (translog != null) {
             translog.toXContent(builder, params);
         }
+        if (suggest != null) {
+            suggest.toXContent(builder, params);
+        }
         return builder;
     }
 }

+ 2 - 1
src/main/java/org/elasticsearch/action/admin/indices/stats/CommonStatsFlags.java

@@ -223,7 +223,8 @@ public class CommonStatsFlags implements Streamable, Cloneable {
         Percolate("percolate"),
         Completion("completion"),
         Segments("segments"),
-        Translog("translog");
+        Translog("translog"),
+        Suggest("suggest");
 
         private final String restName;
 

+ 9 - 0
src/main/java/org/elasticsearch/action/admin/indices/stats/IndicesStatsRequest.java

@@ -247,6 +247,15 @@ public class IndicesStatsRequest extends BroadcastOperationRequest<IndicesStatsR
         return flags.isSet(Flag.Translog);
     }
 
+    public IndicesStatsRequest suggest(boolean suggest) {
+        flags.set(Flag.Suggest, suggest);
+        return this;
+    }
+
+    public boolean suggest() {
+        return flags.isSet(Flag.Suggest);
+    }
+
     @Override
     public void writeTo(StreamOutput out) throws IOException {
         super.writeTo(out);

+ 5 - 0
src/main/java/org/elasticsearch/action/admin/indices/stats/IndicesStatsRequestBuilder.java

@@ -159,6 +159,11 @@ public class IndicesStatsRequestBuilder extends BroadcastOperationRequestBuilder
         return this;
     }
 
+    public IndicesStatsRequestBuilder setSuggest(boolean suggest) {
+        request.suggest(suggest);
+        return this;
+    }
+
     @Override
     protected void doExecute(ActionListener<IndicesStatsResponse> listener) {
         ((IndicesAdminClient) client).stats(request, listener);

+ 3 - 0
src/main/java/org/elasticsearch/action/admin/indices/stats/TransportIndicesStatsAction.java

@@ -193,6 +193,9 @@ public class TransportIndicesStatsAction extends TransportBroadcastOperationActi
         if (request.request.translog()) {
             flags.set(CommonStatsFlags.Flag.Translog);
         }
+        if (request.request.suggest()) {
+            flags.set(CommonStatsFlags.Flag.Suggest);
+        }
 
         return new ShardStats(indexShard, flags);
     }

+ 5 - 0
src/main/java/org/elasticsearch/action/suggest/TransportSuggestAction.java

@@ -39,6 +39,7 @@ import org.elasticsearch.common.xcontent.XContentParser;
 import org.elasticsearch.index.engine.Engine;
 import org.elasticsearch.index.service.IndexService;
 import org.elasticsearch.index.shard.service.IndexShard;
+import org.elasticsearch.index.suggest.stats.ShardSuggestService;
 import org.elasticsearch.indices.IndicesService;
 import org.elasticsearch.search.suggest.Suggest;
 import org.elasticsearch.search.suggest.SuggestPhase;
@@ -150,6 +151,9 @@ public class TransportSuggestAction extends TransportBroadcastOperationAction<Su
         IndexService indexService = indicesService.indexServiceSafe(request.index());
         IndexShard indexShard = indexService.shardSafe(request.shardId());
         final Engine.Searcher searcher = indexShard.acquireSearcher("suggest");
+        ShardSuggestService shardSuggestService = indexShard.shardSuggestService();
+        shardSuggestService.preSuggest();
+        long startTime = System.nanoTime();
         XContentParser parser = null;
         try {
             BytesReference suggest = request.suggest();
@@ -170,6 +174,7 @@ public class TransportSuggestAction extends TransportBroadcastOperationAction<Su
             if (parser != null) {
                 parser.close();
             }
+            shardSuggestService.postSuggest(System.nanoTime() - startTime);
         }
     }
 }

+ 2 - 0
src/main/java/org/elasticsearch/index/service/InternalIndexService.java

@@ -65,6 +65,7 @@ import org.elasticsearch.index.snapshots.IndexShardSnapshotModule;
 import org.elasticsearch.index.store.IndexStore;
 import org.elasticsearch.index.store.Store;
 import org.elasticsearch.index.store.StoreModule;
+import org.elasticsearch.index.suggest.SuggestShardModule;
 import org.elasticsearch.index.termvectors.ShardTermVectorModule;
 import org.elasticsearch.index.translog.Translog;
 import org.elasticsearch.index.translog.TranslogModule;
@@ -334,6 +335,7 @@ public class InternalIndexService extends AbstractIndexComponent implements Inde
         modules.add(new PercolatorShardModule());
         modules.add(new ShardTermVectorModule());
         modules.add(new IndexShardSnapshotModule());
+        modules.add(new SuggestShardModule());
 
         Injector shardInjector;
         try {

+ 6 - 0
src/main/java/org/elasticsearch/index/shard/service/IndexShard.java

@@ -53,6 +53,8 @@ import org.elasticsearch.index.shard.IllegalIndexShardStateException;
 import org.elasticsearch.index.shard.IndexShardComponent;
 import org.elasticsearch.index.shard.IndexShardState;
 import org.elasticsearch.index.store.StoreStats;
+import org.elasticsearch.index.suggest.stats.ShardSuggestService;
+import org.elasticsearch.index.suggest.stats.SuggestStats;
 import org.elasticsearch.index.termvectors.ShardTermVectorService;
 import org.elasticsearch.index.translog.TranslogStats;
 import org.elasticsearch.index.warmer.ShardIndexWarmerService;
@@ -108,12 +110,16 @@ public interface IndexShard extends IndexShardComponent {
 
     TranslogStats translogStats();
 
+    SuggestStats suggestStats();
+
     PercolatorQueriesRegistry percolateRegistry();
 
     ShardPercolateService shardPercolateService();
 
     ShardTermVectorService termVectorService();
 
+    ShardSuggestService shardSuggestService();
+
     MapperService mapperService();
 
     IndexFieldDataService indexFieldDataService();

+ 15 - 1
src/main/java/org/elasticsearch/index/shard/service/InternalIndexShard.java

@@ -74,6 +74,8 @@ import org.elasticsearch.index.settings.IndexSettingsService;
 import org.elasticsearch.index.shard.*;
 import org.elasticsearch.index.store.Store;
 import org.elasticsearch.index.store.StoreStats;
+import org.elasticsearch.index.suggest.stats.ShardSuggestService;
+import org.elasticsearch.index.suggest.stats.SuggestStats;
 import org.elasticsearch.index.termvectors.ShardTermVectorService;
 import org.elasticsearch.index.translog.Translog;
 import org.elasticsearch.index.translog.TranslogStats;
@@ -122,6 +124,7 @@ public class InternalIndexShard extends AbstractIndexShardComponent implements I
     private final ShardTermVectorService termVectorService;
     private final IndexFieldDataService indexFieldDataService;
     private final IndexService indexService;
+    private final ShardSuggestService shardSuggestService;
 
     private final Object mutex = new Object();
     private final String checkIndexOnStartup;
@@ -146,7 +149,7 @@ public class InternalIndexShard extends AbstractIndexShardComponent implements I
     public InternalIndexShard(ShardId shardId, @IndexSettings Settings indexSettings, IndexSettingsService indexSettingsService, IndicesLifecycle indicesLifecycle, Store store, Engine engine, MergeSchedulerProvider mergeScheduler, Translog translog,
                               ThreadPool threadPool, MapperService mapperService, IndexQueryParserService queryParserService, IndexCache indexCache, IndexAliasesService indexAliasesService, ShardIndexingService indexingService, ShardGetService getService, ShardSearchService searchService, ShardIndexWarmerService shardWarmerService,
                               ShardFilterCache shardFilterCache, ShardFieldData shardFieldData, PercolatorQueriesRegistry percolatorQueriesRegistry, ShardPercolateService shardPercolateService, CodecService codecService,
-                              ShardTermVectorService termVectorService, IndexFieldDataService indexFieldDataService, IndexService indexService) {
+                              ShardTermVectorService termVectorService, IndexFieldDataService indexFieldDataService, IndexService indexService, ShardSuggestService shardSuggestService) {
         super(shardId, indexSettings);
         this.indicesLifecycle = (InternalIndicesLifecycle) indicesLifecycle;
         this.indexSettingsService = indexSettingsService;
@@ -171,6 +174,7 @@ public class InternalIndexShard extends AbstractIndexShardComponent implements I
         this.indexFieldDataService = indexFieldDataService;
         this.indexService = indexService;
         this.codecService = codecService;
+        this.shardSuggestService = shardSuggestService;
         state = IndexShardState.CREATED;
 
         this.refreshInterval = indexSettings.getAsTime(INDEX_REFRESH_INTERVAL, engine.defaultRefreshInterval());
@@ -213,6 +217,11 @@ public class InternalIndexShard extends AbstractIndexShardComponent implements I
         return termVectorService;
     }
 
+    @Override
+    public ShardSuggestService shardSuggestService() {
+        return shardSuggestService;
+    }
+
     @Override
     public IndexFieldDataService indexFieldDataService() {
         return indexFieldDataService;
@@ -560,6 +569,11 @@ public class InternalIndexShard extends AbstractIndexShardComponent implements I
         return translog.stats();
     }
 
+    @Override
+    public SuggestStats suggestStats() {
+        return shardSuggestService.stats();
+    }
+
     @Override
     public CompletionStats completionStats(String... fields) {
         CompletionStats completionStats = new CompletionStats();

+ 33 - 0
src/main/java/org/elasticsearch/index/suggest/SuggestShardModule.java

@@ -0,0 +1,33 @@
+/*
+ * Licensed to Elasticsearch under one or more contributor
+ * license agreements. See the NOTICE file distributed with
+ * this work for additional information regarding copyright
+ * ownership. Elasticsearch 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.index.suggest;
+
+import org.elasticsearch.common.inject.AbstractModule;
+import org.elasticsearch.index.suggest.stats.ShardSuggestService;
+
+/**
+ *
+ */
+public class SuggestShardModule extends AbstractModule {
+    @Override
+    protected void configure() {
+        bind(ShardSuggestService.class).asEagerSingleton();
+    }
+}

+ 68 - 0
src/main/java/org/elasticsearch/index/suggest/stats/ShardSuggestService.java

@@ -0,0 +1,68 @@
+/*
+ * Licensed to Elasticsearch under one or more contributor
+ * license agreements. See the NOTICE file distributed with
+ * this work for additional information regarding copyright
+ * ownership. Elasticsearch 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.index.suggest.stats;
+
+import org.elasticsearch.common.inject.Inject;
+import org.elasticsearch.common.metrics.CounterMetric;
+import org.elasticsearch.common.metrics.MeanMetric;
+import org.elasticsearch.common.settings.Settings;
+import org.elasticsearch.index.settings.IndexSettings;
+import org.elasticsearch.index.shard.AbstractIndexShardComponent;
+import org.elasticsearch.index.shard.ShardId;
+
+import java.util.concurrent.TimeUnit;
+
+/**
+ *
+ */
+public class ShardSuggestService extends AbstractIndexShardComponent {
+
+    private final MeanMetric suggestMetric = new MeanMetric();
+    private final CounterMetric currentMetric = new CounterMetric();
+
+    @Inject
+    public ShardSuggestService(ShardId shardId, @IndexSettings Settings indexSettings) {
+        super(shardId, indexSettings);
+    }
+
+    /**
+     * Called before suggest
+     */
+    public void preSuggest() {
+        currentMetric.inc();
+    }
+
+    /**
+     * Called after suggest
+     * @param tookInNanos time of suggest used in nanos
+     */
+    public void postSuggest(long tookInNanos) {
+        currentMetric.dec();
+        suggestMetric.inc(tookInNanos);
+    }
+
+    /**
+     * @return The current stats
+     */
+    public SuggestStats stats() {
+        return new SuggestStats(suggestMetric.count(), TimeUnit.NANOSECONDS.toMillis(suggestMetric.sum()), currentMetric.count());
+    }
+
+}

+ 124 - 0
src/main/java/org/elasticsearch/index/suggest/stats/SuggestStats.java

@@ -0,0 +1,124 @@
+/*
+ * Licensed to Elasticsearch under one or more contributor
+ * license agreements. See the NOTICE file distributed with
+ * this work for additional information regarding copyright
+ * ownership. Elasticsearch 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.index.suggest.stats;
+
+import org.elasticsearch.common.io.stream.StreamInput;
+import org.elasticsearch.common.io.stream.StreamOutput;
+import org.elasticsearch.common.io.stream.Streamable;
+import org.elasticsearch.common.unit.TimeValue;
+import org.elasticsearch.common.xcontent.ToXContent;
+import org.elasticsearch.common.xcontent.XContentBuilder;
+import org.elasticsearch.common.xcontent.XContentBuilderString;
+
+import java.io.IOException;
+
+/**
+ * Exposes suggest related statistics.
+ */
+public class SuggestStats implements Streamable, ToXContent {
+
+    private long suggestCount;
+    private long suggestTimeInMillis;
+    private long current;
+
+    public SuggestStats() {
+    }
+
+    SuggestStats(long suggestCount, long suggestTimeInMillis, long current) {
+        this.suggestCount = suggestCount;
+        this.suggestTimeInMillis = suggestTimeInMillis;
+        this.current = current;
+    }
+
+    /**
+     * @return The number of times the suggest api has been invoked.
+     */
+    public long getCount() {
+        return suggestCount;
+    }
+
+    /**
+     * @return The total amount of time spend in the suggest api
+     */
+    public long getTimeInMillis() {
+        return suggestTimeInMillis;
+    }
+
+    /**
+     * @return The total amount of time spend in the suggest api
+     */
+    public TimeValue getTime() {
+        return new TimeValue(getTimeInMillis());
+    }
+
+    /**
+     * @return The total amount of active suggest api invocations.
+     */
+    public long getCurrent() {
+        return current;
+    }
+
+    public void add(SuggestStats suggestStats) {
+        if (suggestStats != null) {
+            suggestCount += suggestStats.getCount();
+            suggestTimeInMillis += suggestStats.getTimeInMillis();
+            current += suggestStats.getCurrent();
+        }
+    }
+
+    public static SuggestStats readSuggestStats(StreamInput in) throws IOException {
+        SuggestStats stats = new SuggestStats();
+        stats.readFrom(in);
+        return stats;
+    }
+
+    static final class Fields {
+        static final XContentBuilderString SUGGEST = new XContentBuilderString("suggest");
+        static final XContentBuilderString TOTAL = new XContentBuilderString("total");
+        static final XContentBuilderString TIME = new XContentBuilderString("time");
+        static final XContentBuilderString TIME_IN_MILLIS = new XContentBuilderString("time_in_millis");
+        static final XContentBuilderString CURRENT = new XContentBuilderString("current");
+    }
+
+
+    @Override
+    public void readFrom(StreamInput in) throws IOException {
+        suggestCount = in.readVLong();
+        suggestTimeInMillis = in.readVLong();
+        current = in.readVLong();
+    }
+
+    @Override
+    public void writeTo(StreamOutput out) throws IOException {
+        out.writeVLong(suggestCount);
+        out.writeVLong(suggestTimeInMillis);
+        out.writeVLong(current);
+    }
+
+    @Override
+    public XContentBuilder toXContent(XContentBuilder builder, Params params) throws IOException {
+        builder.startObject(Fields.SUGGEST);
+        builder.field(Fields.TOTAL, suggestCount);
+        builder.timeValueField(Fields.TIME_IN_MILLIS, Fields.TIME, suggestTimeInMillis);
+        builder.field(Fields.CURRENT, current);
+        builder.endObject();
+        return builder;
+    }
+}

+ 6 - 0
src/main/java/org/elasticsearch/indices/NodeIndicesStats.java

@@ -47,6 +47,7 @@ import org.elasticsearch.index.shard.DocsStats;
 import org.elasticsearch.index.shard.ShardId;
 import org.elasticsearch.index.shard.service.IndexShard;
 import org.elasticsearch.index.store.StoreStats;
+import org.elasticsearch.index.suggest.stats.SuggestStats;
 import org.elasticsearch.search.suggest.completion.CompletionStats;
 
 import java.io.IOException;
@@ -150,6 +151,11 @@ public class NodeIndicesStats implements Streamable, Serializable, ToXContent {
         return stats.getSegments();
     }
 
+    @Nullable
+    public SuggestStats getSuggest() {
+        return stats.getSuggest();
+    }
+
     public static NodeIndicesStats readIndicesStats(StreamInput in) throws IOException {
         NodeIndicesStats stats = new NodeIndicesStats();
         stats.readFrom(in);

+ 1 - 0
src/main/java/org/elasticsearch/rest/action/admin/indices/stats/RestIndicesStatsAction.java

@@ -89,6 +89,7 @@ public class RestIndicesStatsAction extends BaseRestHandler {
             indicesStatsRequest.segments(metrics.contains("segments"));
             indicesStatsRequest.fieldData(metrics.contains("fielddata"));
             indicesStatsRequest.completion(metrics.contains("completion"));
+            indicesStatsRequest.suggest(metrics.contains("suggest"));
         }
 
         if (indicesStatsRequest.completion() && (request.hasParam("fields") || request.hasParam("completion_fields"))) {

+ 19 - 0
src/main/java/org/elasticsearch/rest/action/cat/RestIndicesAction.java

@@ -277,6 +277,16 @@ public class RestIndicesAction extends AbstractCatAction {
         table.addCell("warmer.total_time", "sibling:pri;alias:wtt,warmerTotalTime;default:false;text-align:right;desc:time spent in warmers");
         table.addCell("pri.warmer.total_time", "default:false;text-align:right;desc:time spent in warmers");
 
+        table.addCell("suggest.current", "sibling:pri;alias:suc,suggestCurrent;default:false;text-align:right;desc:number of current suggest ops");
+        table.addCell("pri.suggest.current", "default:false;text-align:right;desc:number of current suggest ops");
+
+        table.addCell("suggest.time", "sibling:pri;alias:suti,suggestTime;default:false;text-align:right;desc:time spend in suggest");
+        table.addCell("pri.suggest.time", "default:false;text-align:right;desc:time spend in suggest");
+
+        table.addCell("suggest.total", "sibling:pri;alias:suto,suggestTotal;default:false;text-align:right;desc:number of suggest ops");
+        table.addCell("pri.suggest.total", "default:false;text-align:right;desc:number of suggest ops");
+
+
         table.endHeaders();
         return table;
     }
@@ -440,6 +450,15 @@ public class RestIndicesAction extends AbstractCatAction {
             table.addCell(indexStats == null ? null : indexStats.getTotal().getWarmer().totalTime());
             table.addCell(indexStats == null ? null : indexStats.getPrimaries().getWarmer().totalTime());
 
+            table.addCell(indexStats == null ? null : indexStats.getTotal().getSuggest().getCurrent());
+            table.addCell(indexStats == null ? null : indexStats.getPrimaries().getSuggest().getCurrent());
+
+            table.addCell(indexStats == null ? null : indexStats.getTotal().getSuggest().getTime());
+            table.addCell(indexStats == null ? null : indexStats.getPrimaries().getSuggest().getTime());
+
+            table.addCell(indexStats == null ? null : indexStats.getTotal().getSuggest().getCount());
+            table.addCell(indexStats == null ? null : indexStats.getPrimaries().getSuggest().getCount());
+
             table.endRow();
         }
 

+ 8 - 0
src/main/java/org/elasticsearch/rest/action/cat/RestNodesAction.java

@@ -201,6 +201,10 @@ public class RestNodesAction extends AbstractCatAction {
         table.addCell("segments.count", "alias:sc,segmentsCount;default:false;text-align:right;desc:number of segments");
         table.addCell("segments.memory", "alias:sm,segmentsMemory;default:false;text-align:right;desc:memory used by segments");
 
+        table.addCell("suggest.current", "alias:suc,suggestCurrent;default:false;text-align:right;desc:number of current suggest ops");
+        table.addCell("suggest.time", "alias:suti,suggestTime;default:false;text-align:right;desc:time spend in suggest");
+        table.addCell("suggest.total", "alias:suto,suggestTotal;default:false;text-align:right;desc:number of suggest ops");
+
         table.endHeaders();
         return table;
     }
@@ -299,6 +303,10 @@ public class RestNodesAction extends AbstractCatAction {
             table.addCell(stats == null ? null : stats.getIndices().getSegments().getCount());
             table.addCell(stats == null ? null : stats.getIndices().getSegments().getMemory());
 
+            table.addCell(stats == null ? null : stats.getIndices().getSuggest().getCurrent());
+            table.addCell(stats == null ? null : stats.getIndices().getSuggest().getTime());
+            table.addCell(stats == null ? null : stats.getIndices().getSuggest().getCount());
+
             table.endRow();
         }
 

+ 145 - 0
src/test/java/org/elasticsearch/index/suggest/stats/SuggestStatsTests.java

@@ -0,0 +1,145 @@
+/*
+ * Licensed to Elasticsearch under one or more contributor
+ * license agreements. See the NOTICE file distributed with
+ * this work for additional information regarding copyright
+ * ownership. Elasticsearch 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.index.suggest.stats;
+
+import org.elasticsearch.action.admin.cluster.node.stats.NodeStats;
+import org.elasticsearch.action.admin.cluster.node.stats.NodesStatsResponse;
+import org.elasticsearch.action.admin.indices.stats.IndicesStatsResponse;
+import org.elasticsearch.action.suggest.SuggestResponse;
+import org.elasticsearch.cluster.ClusterState;
+import org.elasticsearch.cluster.routing.GroupShardsIterator;
+import org.elasticsearch.cluster.routing.ShardIterator;
+import org.elasticsearch.cluster.routing.ShardRouting;
+import org.elasticsearch.common.settings.ImmutableSettings;
+import org.elasticsearch.test.ElasticsearchIntegrationTest;
+import org.junit.Test;
+
+import java.util.HashSet;
+import java.util.Set;
+
+import static org.elasticsearch.cluster.metadata.IndexMetaData.SETTING_NUMBER_OF_REPLICAS;
+import static org.elasticsearch.cluster.metadata.IndexMetaData.SETTING_NUMBER_OF_SHARDS;
+import static org.elasticsearch.test.hamcrest.ElasticsearchAssertions.assertAcked;
+import static org.elasticsearch.test.hamcrest.ElasticsearchAssertions.assertAllSuccessful;
+import static org.hamcrest.Matchers.*;
+
+/**
+ */
+public class SuggestStatsTests extends ElasticsearchIntegrationTest {
+
+    @Override
+    protected int numberOfReplicas() {
+        return 0;
+    }
+
+    @Test
+    public void testSimpleStats() throws Exception {
+        // clear all stats first
+        client().admin().indices().prepareStats().clear().execute().actionGet();
+        final int numNodes = cluster().dataNodes();
+        assertThat(numNodes, greaterThanOrEqualTo(2));
+        final int shardsIdx1 = randomIntBetween(1, 10); // we make sure each node gets at least a single shard...
+        final int shardsIdx2 = Math.max(numNodes - shardsIdx1, randomIntBetween(1, 10));
+        final int totalShards = shardsIdx1 + shardsIdx2;
+        assertThat(numNodes, lessThanOrEqualTo(totalShards));
+        assertAcked(prepareCreate("test1").setSettings(ImmutableSettings.builder()
+                .put(SETTING_NUMBER_OF_SHARDS, shardsIdx1)
+                .put(SETTING_NUMBER_OF_REPLICAS, 0)));
+        assertAcked(prepareCreate("test2").setSettings(ImmutableSettings.builder()
+                .put(SETTING_NUMBER_OF_SHARDS, shardsIdx2)
+                .put(SETTING_NUMBER_OF_REPLICAS, 0)));
+        assertThat(shardsIdx1+shardsIdx2, equalTo(numAssignedShards("test1", "test2")));
+        assertThat(numAssignedShards("test1", "test2"), greaterThanOrEqualTo(2));
+        ensureGreen();
+        int suggestAllIdx = scaledRandomIntBetween(20, 50);
+        int suggestIdx1 = scaledRandomIntBetween(20, 50);
+        int suggestIdx2 = scaledRandomIntBetween(20, 50);
+
+        long startTime = System.currentTimeMillis();
+        for (int i = 0; i < suggestAllIdx; i++) {
+            SuggestResponse suggestResponse = cluster().clientNodeClient().prepareSuggest().setSuggestText("test").get();
+            assertAllSuccessful(suggestResponse);
+        }
+        for (int i = 0; i < suggestIdx1; i++) {
+            SuggestResponse suggestResponse = cluster().clientNodeClient().prepareSuggest("test1").setSuggestText("test").get();
+            assertAllSuccessful(suggestResponse);
+        }
+        for (int i = 0; i < suggestIdx2; i++) {
+            SuggestResponse suggestResponse = cluster().clientNodeClient().prepareSuggest("test2").setSuggestText("test").get();
+            assertAllSuccessful(suggestResponse);
+        }
+        long endTime = System.currentTimeMillis();
+
+        IndicesStatsResponse indicesStats = client().admin().indices().prepareStats().execute().actionGet();
+
+        // check current
+        assertThat(indicesStats.getTotal().getSuggest().getCurrent(), equalTo(0l));
+
+        // check suggest count
+        assertThat(indicesStats.getTotal().getSuggest().getCount(), equalTo((long)(suggestAllIdx * totalShards + suggestIdx1 * shardsIdx1 + suggestIdx2 * shardsIdx2)));
+        assertThat(indicesStats.getIndices().get("test1").getTotal().getSuggest().getCount(), equalTo((long)((suggestAllIdx + suggestIdx1) * shardsIdx1)));
+        assertThat(indicesStats.getIndices().get("test2").getTotal().getSuggest().getCount(), equalTo((long)((suggestAllIdx + suggestIdx2) * shardsIdx2)));
+
+        // check suggest time
+        assertThat(indicesStats.getTotal().getSuggest().getTimeInMillis(), greaterThan(0l));
+        assertThat(indicesStats.getTotal().getSuggest().getTimeInMillis(), lessThanOrEqualTo(endTime - startTime));
+
+        NodesStatsResponse nodeStats = client().admin().cluster().prepareNodesStats().execute().actionGet();
+        NodeStats[] nodes = nodeStats.getNodes();
+        Set<String> nodeIdsWithIndex = nodeIdsWithIndex("test1", "test2");
+        int num = 0;
+        for (NodeStats stat : nodes) {
+            SuggestStats suggestStats = stat.getIndices().getSuggest();
+            if (nodeIdsWithIndex.contains(stat.getNode().getId())) {
+                assertThat(suggestStats.getCount(), greaterThan(0l));
+                assertThat(suggestStats.getTimeInMillis(), greaterThan(0l));
+                num++;
+            } else {
+                assertThat(suggestStats.getCount(), equalTo(0l));
+                assertThat(suggestStats.getTimeInMillis(), equalTo(0l));
+            }
+        }
+
+        assertThat(num, greaterThan(0));
+     
+    }
+
+    private Set<String> nodeIdsWithIndex(String... indices) {
+        ClusterState state = client().admin().cluster().prepareState().execute().actionGet().getState();
+        GroupShardsIterator allAssignedShardsGrouped = state.routingTable().allAssignedShardsGrouped(indices, true);
+        Set<String> nodes = new HashSet<String>();
+        for (ShardIterator shardIterator : allAssignedShardsGrouped) {
+            for (ShardRouting routing : shardIterator.asUnordered()) {
+                if (routing.active()) {
+                    nodes.add(routing.currentNodeId());
+                }
+
+            }
+        }
+        return nodes;
+    }
+
+
+    protected int numAssignedShards(String... indices) {
+        ClusterState state = client().admin().cluster().prepareState().execute().actionGet().getState();
+        GroupShardsIterator allAssignedShardsGrouped = state.routingTable().allAssignedShardsGrouped(indices, true);
+        return allAssignedShardsGrouped.size();
+    }
+}

+ 6 - 1
src/test/java/org/elasticsearch/indices/stats/SimpleIndexStatsTests.java

@@ -315,7 +315,7 @@ public class SimpleIndexStatsTests extends ElasticsearchIntegrationTest {
     @Test
     public void testFlagOrdinalOrder() {
         Flag[] flags = new Flag[]{Flag.Store, Flag.Indexing, Flag.Get, Flag.Search, Flag.Merge, Flag.Flush, Flag.Refresh,
-                Flag.FilterCache, Flag.IdCache, Flag.FieldData, Flag.Docs, Flag.Warmer, Flag.Percolate, Flag.Completion, Flag.Segments, Flag.Translog};
+                Flag.FilterCache, Flag.IdCache, Flag.FieldData, Flag.Docs, Flag.Warmer, Flag.Percolate, Flag.Completion, Flag.Segments, Flag.Translog, Flag.Suggest};
 
         assertThat(flags.length, equalTo(Flag.values().length));
         for (int i = 0; i < flags.length; i++) {
@@ -373,6 +373,9 @@ public class SimpleIndexStatsTests extends ElasticsearchIntegrationTest {
             case Translog:
                 builder.setTranslog(set);
                 break;
+            case Suggest:
+                builder.setSuggest(set);
+                break;
             default:
                 fail("new flag? " + flag);
                 break;
@@ -413,6 +416,8 @@ public class SimpleIndexStatsTests extends ElasticsearchIntegrationTest {
                 return response.getSegments() != null;
             case Translog:
                 return response.getTranslog() != null;
+            case Suggest:
+                return response.getSuggest() != null;
             default:
                 fail("new flag? " + flag);
                 return false;