Skip to content

Commit

Permalink
Merge remote-tracking branch 'es/master' into ccr
Browse files Browse the repository at this point in the history
* es/master: (24 commits)
  Add missing_bucket option in the composite agg (#29465)
  Rename index_prefix to index_prefixes (#30932)
  Rename methods in PersistentTasksService (#30837)
  [DOCS] Fix watcher file location
  Update the version checks around range bucket keys, now that the change was backported.
  Use dedicated ML APIs in tests (#30941)
  [DOCS] Remove reference to platinum Docker image (#30916)
  Minor clean-up in InternalRange. (#30886)
  stable filemode for zip distributions (#30854)
  [DOCS] Adds missing TLS settings for auditing (#30822)
  [test] packaging: use shell when running commands (#30852)
  Fix location of AbstractHttpServerTransport (#30888)
  [test] packaging test logging for suse distros
  Moved keyword tokenizer to analysis-common module (#30642)
  Upgrade to Lucene-7.4-snapshot-1cbadda4d3 (#30928)
  Limit the scope of BouncyCastle dependency (#30358)
  [DOCS] Reset edit links (#30909)
  Fix IndexTemplateMetaData parsing from xContent (#30917)
  Remove log traces in AzureStorageServiceImpl and fix test (#30924)
  Deprecate accepting malformed requests in stored script API (#28939)
  ...
  • Loading branch information
martijnvg committed May 30, 2018
2 parents 03e3bd2 + e33d107 commit 56472d6
Show file tree
Hide file tree
Showing 409 changed files with 7,656 additions and 2,631 deletions.
2 changes: 1 addition & 1 deletion buildSrc/version.properties
Original file line number Diff line number Diff line change
@@ -1,5 +1,5 @@
elasticsearch = 7.0.0-alpha1
lucene = 7.4.0-snapshot-cc2ee23050
lucene = 7.4.0-snapshot-1cbadda4d3

# optional dependencies
spatial4j = 0.7
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -21,8 +21,6 @@

import org.apache.http.Header;
import org.elasticsearch.action.ActionListener;
import org.elasticsearch.action.admin.cluster.node.tasks.list.ListTasksRequest;
import org.elasticsearch.action.admin.cluster.node.tasks.list.ListTasksResponse;
import org.elasticsearch.action.admin.cluster.settings.ClusterUpdateSettingsRequest;
import org.elasticsearch.action.admin.cluster.settings.ClusterUpdateSettingsResponse;
import org.elasticsearch.action.ingest.PutPipelineRequest;
Expand Down Expand Up @@ -68,28 +66,6 @@ public void putSettingsAsync(ClusterUpdateSettingsRequest clusterUpdateSettingsR
ClusterUpdateSettingsResponse::fromXContent, listener, emptySet(), headers);
}

/**
* Get current tasks using the Task Management API
* <p>
* See
* <a href="https://www.elastic.co/guide/en/elasticsearch/reference/current/tasks.html"> Task Management API on elastic.co</a>
*/
public ListTasksResponse listTasks(ListTasksRequest request, Header... headers) throws IOException {
return restHighLevelClient.performRequestAndParseEntity(request, RequestConverters::listTasks, ListTasksResponse::fromXContent,
emptySet(), headers);
}

/**
* Asynchronously get current tasks using the Task Management API
* <p>
* See
* <a href="https://www.elastic.co/guide/en/elasticsearch/reference/current/tasks.html"> Task Management API on elastic.co</a>
*/
public void listTasksAsync(ListTasksRequest request, ActionListener<ListTasksResponse> listener, Header... headers) {
restHighLevelClient.performRequestAsyncAndParseEntity(request, RequestConverters::listTasks, ListTasksResponse::fromXContent,
listener, emptySet(), headers);
}

/**
* Add a pipeline or update an existing pipeline in the cluster
* <p>
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -192,6 +192,7 @@ public class RestHighLevelClient implements Closeable {
private final IndicesClient indicesClient = new IndicesClient(this);
private final ClusterClient clusterClient = new ClusterClient(this);
private final SnapshotClient snapshotClient = new SnapshotClient(this);
private final TasksClient tasksClient = new TasksClient(this);

/**
* Creates a {@link RestHighLevelClient} given the low level {@link RestClientBuilder} that allows to build the
Expand Down Expand Up @@ -264,6 +265,15 @@ public final SnapshotClient snapshot() {
return snapshotClient;
}

/**
* Provides a {@link TasksClient} which can be used to access the Tasks API.
*
* See <a href="https://www.elastic.co/guide/en/elasticsearch/reference/current/tasks.html">Task Management API on elastic.co</a>
*/
public final TasksClient tasks() {
return tasksClient;
}

/**
* Executes a bulk request using the Bulk API
*
Expand Down
Original file line number Diff line number Diff line change
@@ -0,0 +1,64 @@
/*
* 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.client;

import org.apache.http.Header;
import org.elasticsearch.action.ActionListener;
import org.elasticsearch.action.admin.cluster.node.tasks.list.ListTasksRequest;
import org.elasticsearch.action.admin.cluster.node.tasks.list.ListTasksResponse;

import java.io.IOException;

import static java.util.Collections.emptySet;

/**
* A wrapper for the {@link RestHighLevelClient} that provides methods for accessing the Tasks API.
* <p>
* See <a href="https://www.elastic.co/guide/en/elasticsearch/reference/current/tasks.html">Task Management API on elastic.co</a>
*/
public class TasksClient {
private final RestHighLevelClient restHighLevelClient;

TasksClient(RestHighLevelClient restHighLevelClient) {
this.restHighLevelClient = restHighLevelClient;
}

/**
* Get current tasks using the Task Management API
* <p>
* See
* <a href="https://www.elastic.co/guide/en/elasticsearch/reference/current/tasks.html"> Task Management API on elastic.co</a>
*/
public ListTasksResponse list(ListTasksRequest request, Header... headers) throws IOException {
return restHighLevelClient.performRequestAndParseEntity(request, RequestConverters::listTasks, ListTasksResponse::fromXContent,
emptySet(), headers);
}

/**
* Asynchronously get current tasks using the Task Management API
* <p>
* See
* <a href="https://www.elastic.co/guide/en/elasticsearch/reference/current/tasks.html"> Task Management API on elastic.co</a>
*/
public void listAsync(ListTasksRequest request, ActionListener<ListTasksResponse> listener, Header... headers) {
restHighLevelClient.performRequestAsyncAndParseEntity(request, RequestConverters::listTasks, ListTasksResponse::fromXContent,
listener, emptySet(), headers);
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -20,9 +20,6 @@
package org.elasticsearch.client;

import org.elasticsearch.ElasticsearchException;
import org.elasticsearch.action.admin.cluster.node.tasks.list.ListTasksRequest;
import org.elasticsearch.action.admin.cluster.node.tasks.list.ListTasksResponse;
import org.elasticsearch.action.admin.cluster.node.tasks.list.TaskGroup;
import org.elasticsearch.action.admin.cluster.settings.ClusterUpdateSettingsRequest;
import org.elasticsearch.action.admin.cluster.settings.ClusterUpdateSettingsResponse;
import org.elasticsearch.action.ingest.PutPipelineRequest;
Expand All @@ -37,16 +34,13 @@
import org.elasticsearch.indices.recovery.RecoverySettings;
import org.elasticsearch.ingest.Pipeline;
import org.elasticsearch.rest.RestStatus;
import org.elasticsearch.tasks.TaskInfo;

import java.io.IOException;
import java.util.HashMap;
import java.util.Map;

import static java.util.Collections.emptyList;
import static org.elasticsearch.test.hamcrest.ElasticsearchAssertions.assertAcked;
import static org.hamcrest.Matchers.equalTo;
import static org.hamcrest.Matchers.greaterThanOrEqualTo;
import static org.hamcrest.Matchers.notNullValue;
import static org.hamcrest.Matchers.nullValue;

Expand Down Expand Up @@ -117,31 +111,6 @@ public void testClusterUpdateSettingNonExistent() {
"Elasticsearch exception [type=illegal_argument_exception, reason=transient setting [" + setting + "], not recognized]"));
}

public void testListTasks() throws IOException {
ListTasksRequest request = new ListTasksRequest();
ListTasksResponse response = execute(request, highLevelClient().cluster()::listTasks, highLevelClient().cluster()::listTasksAsync);

assertThat(response, notNullValue());
assertThat(response.getNodeFailures(), equalTo(emptyList()));
assertThat(response.getTaskFailures(), equalTo(emptyList()));
// It's possible that there are other tasks except 'cluster:monitor/tasks/lists[n]' and 'action":"cluster:monitor/tasks/lists'
assertThat(response.getTasks().size(), greaterThanOrEqualTo(2));
boolean listTasksFound = false;
for (TaskGroup taskGroup : response.getTaskGroups()) {
TaskInfo parent = taskGroup.getTaskInfo();
if ("cluster:monitor/tasks/lists".equals(parent.getAction())) {
assertThat(taskGroup.getChildTasks().size(), equalTo(1));
TaskGroup childGroup = taskGroup.getChildTasks().iterator().next();
assertThat(childGroup.getChildTasks().isEmpty(), equalTo(true));
TaskInfo child = childGroup.getTaskInfo();
assertThat(child.getAction(), equalTo("cluster:monitor/tasks/lists[n]"));
assertThat(child.getParentTaskId(), equalTo(parent.getTaskId()));
listTasksFound = true;
}
}
assertTrue("List tasks were not found", listTasksFound);
}

public void testPutPipeline() throws IOException {
String id = "some_pipeline_id";
XContentType xContentType = randomFrom(XContentType.values());
Expand Down
Original file line number Diff line number Diff line change
@@ -0,0 +1,61 @@
/*
* 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.client;

import org.elasticsearch.action.admin.cluster.node.tasks.list.ListTasksRequest;
import org.elasticsearch.action.admin.cluster.node.tasks.list.ListTasksResponse;
import org.elasticsearch.action.admin.cluster.node.tasks.list.TaskGroup;
import org.elasticsearch.tasks.TaskInfo;

import java.io.IOException;

import static java.util.Collections.emptyList;
import static org.hamcrest.Matchers.equalTo;
import static org.hamcrest.Matchers.greaterThanOrEqualTo;
import static org.hamcrest.Matchers.notNullValue;

public class TasksIT extends ESRestHighLevelClientTestCase {

public void testListTasks() throws IOException {
ListTasksRequest request = new ListTasksRequest();
ListTasksResponse response = execute(request, highLevelClient().tasks()::list, highLevelClient().tasks()::listAsync);

assertThat(response, notNullValue());
assertThat(response.getNodeFailures(), equalTo(emptyList()));
assertThat(response.getTaskFailures(), equalTo(emptyList()));
// It's possible that there are other tasks except 'cluster:monitor/tasks/lists[n]' and 'action":"cluster:monitor/tasks/lists'
assertThat(response.getTasks().size(), greaterThanOrEqualTo(2));
boolean listTasksFound = false;
for (TaskGroup taskGroup : response.getTaskGroups()) {
TaskInfo parent = taskGroup.getTaskInfo();
if ("cluster:monitor/tasks/lists".equals(parent.getAction())) {
assertThat(taskGroup.getChildTasks().size(), equalTo(1));
TaskGroup childGroup = taskGroup.getChildTasks().iterator().next();
assertThat(childGroup.getChildTasks().isEmpty(), equalTo(true));
TaskInfo child = childGroup.getTaskInfo();
assertThat(child.getAction(), equalTo("cluster:monitor/tasks/lists[n]"));
assertThat(child.getParentTaskId(), equalTo(parent.getTaskId()));
listTasksFound = true;
}
}
assertTrue("List tasks were not found", listTasksFound);
}

}
Original file line number Diff line number Diff line change
Expand Up @@ -19,13 +19,8 @@

package org.elasticsearch.client.documentation;

import org.elasticsearch.ElasticsearchException;
import org.elasticsearch.action.ActionListener;
import org.elasticsearch.action.LatchedActionListener;
import org.elasticsearch.action.TaskOperationFailure;
import org.elasticsearch.action.admin.cluster.node.tasks.list.ListTasksRequest;
import org.elasticsearch.action.admin.cluster.node.tasks.list.ListTasksResponse;
import org.elasticsearch.action.admin.cluster.node.tasks.list.TaskGroup;
import org.elasticsearch.action.admin.cluster.settings.ClusterUpdateSettingsRequest;
import org.elasticsearch.action.admin.cluster.settings.ClusterUpdateSettingsResponse;
import org.elasticsearch.action.ingest.PutPipelineRequest;
Expand All @@ -39,21 +34,15 @@
import org.elasticsearch.common.unit.TimeValue;
import org.elasticsearch.common.xcontent.XContentType;
import org.elasticsearch.indices.recovery.RecoverySettings;
import org.elasticsearch.tasks.TaskId;
import org.elasticsearch.tasks.TaskInfo;

import java.io.IOException;
import java.nio.charset.StandardCharsets;
import java.util.HashMap;
import java.util.List;
import java.util.Map;
import java.util.concurrent.CountDownLatch;
import java.util.concurrent.TimeUnit;

import static java.util.Collections.emptyList;
import static org.hamcrest.Matchers.equalTo;
import static org.hamcrest.Matchers.greaterThanOrEqualTo;
import static org.hamcrest.Matchers.notNullValue;

/**
* This class is used to generate the Java Cluster API documentation.
Expand Down Expand Up @@ -193,89 +182,6 @@ public void onFailure(Exception e) {
}
}

public void testListTasks() throws IOException {
RestHighLevelClient client = highLevelClient();
{
// tag::list-tasks-request
ListTasksRequest request = new ListTasksRequest();
// end::list-tasks-request

// tag::list-tasks-request-filter
request.setActions("cluster:*"); // <1>
request.setNodes("nodeId1", "nodeId2"); // <2>
request.setParentTaskId(new TaskId("parentTaskId", 42)); // <3>
// end::list-tasks-request-filter

// tag::list-tasks-request-detailed
request.setDetailed(true); // <1>
// end::list-tasks-request-detailed

// tag::list-tasks-request-wait-completion
request.setWaitForCompletion(true); // <1>
request.setTimeout(TimeValue.timeValueSeconds(50)); // <2>
request.setTimeout("50s"); // <3>
// end::list-tasks-request-wait-completion
}

ListTasksRequest request = new ListTasksRequest();

// tag::list-tasks-execute
ListTasksResponse response = client.cluster().listTasks(request);
// end::list-tasks-execute

assertThat(response, notNullValue());

// tag::list-tasks-response-tasks
List<TaskInfo> tasks = response.getTasks(); // <1>
// end::list-tasks-response-tasks

// tag::list-tasks-response-calc
Map<String, List<TaskInfo>> perNodeTasks = response.getPerNodeTasks(); // <1>
List<TaskGroup> groups = response.getTaskGroups(); // <2>
// end::list-tasks-response-calc

// tag::list-tasks-response-failures
List<ElasticsearchException> nodeFailures = response.getNodeFailures(); // <1>
List<TaskOperationFailure> taskFailures = response.getTaskFailures(); // <2>
// end::list-tasks-response-failures

assertThat(response.getNodeFailures(), equalTo(emptyList()));
assertThat(response.getTaskFailures(), equalTo(emptyList()));
assertThat(response.getTasks().size(), greaterThanOrEqualTo(2));
}

public void testListTasksAsync() throws Exception {
RestHighLevelClient client = highLevelClient();
{
ListTasksRequest request = new ListTasksRequest();

// tag::list-tasks-execute-listener
ActionListener<ListTasksResponse> listener =
new ActionListener<ListTasksResponse>() {
@Override
public void onResponse(ListTasksResponse response) {
// <1>
}

@Override
public void onFailure(Exception e) {
// <2>
}
};
// end::list-tasks-execute-listener

// Replace the empty listener by a blocking listener in test
final CountDownLatch latch = new CountDownLatch(1);
listener = new LatchedActionListener<>(listener, latch);

// tag::list-tasks-execute-async
client.cluster().listTasksAsync(request, listener); // <1>
// end::list-tasks-execute-async

assertTrue(latch.await(30L, TimeUnit.SECONDS));
}
}

public void testPutPipeline() throws IOException {
RestHighLevelClient client = highLevelClient();

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -45,7 +45,7 @@
import static org.hamcrest.Matchers.equalTo;

/**
* This class is used to generate the Java Cluster API documentation.
* This class is used to generate the Java Snapshot API documentation.
* You need to wrap your code between two tags like:
* // tag::example
* // end::example
Expand Down
Loading

0 comments on commit 56472d6

Please sign in to comment.