Indices API: Add open and close index, closes #447.
This commit is contained in:
parent
a76824e395
commit
ca7a7467dc
|
@ -32,12 +32,14 @@ import org.elasticsearch.action.admin.cluster.ping.single.TransportSinglePingAct
|
||||||
import org.elasticsearch.action.admin.cluster.state.TransportClusterStateAction;
|
import org.elasticsearch.action.admin.cluster.state.TransportClusterStateAction;
|
||||||
import org.elasticsearch.action.admin.indices.alias.TransportIndicesAliasesAction;
|
import org.elasticsearch.action.admin.indices.alias.TransportIndicesAliasesAction;
|
||||||
import org.elasticsearch.action.admin.indices.cache.clear.TransportClearIndicesCacheAction;
|
import org.elasticsearch.action.admin.indices.cache.clear.TransportClearIndicesCacheAction;
|
||||||
|
import org.elasticsearch.action.admin.indices.close.TransportCloseIndexAction;
|
||||||
import org.elasticsearch.action.admin.indices.create.TransportCreateIndexAction;
|
import org.elasticsearch.action.admin.indices.create.TransportCreateIndexAction;
|
||||||
import org.elasticsearch.action.admin.indices.delete.TransportDeleteIndexAction;
|
import org.elasticsearch.action.admin.indices.delete.TransportDeleteIndexAction;
|
||||||
import org.elasticsearch.action.admin.indices.flush.TransportFlushAction;
|
import org.elasticsearch.action.admin.indices.flush.TransportFlushAction;
|
||||||
import org.elasticsearch.action.admin.indices.gateway.snapshot.TransportGatewaySnapshotAction;
|
import org.elasticsearch.action.admin.indices.gateway.snapshot.TransportGatewaySnapshotAction;
|
||||||
import org.elasticsearch.action.admin.indices.mapping.delete.TransportDeleteMappingAction;
|
import org.elasticsearch.action.admin.indices.mapping.delete.TransportDeleteMappingAction;
|
||||||
import org.elasticsearch.action.admin.indices.mapping.put.TransportPutMappingAction;
|
import org.elasticsearch.action.admin.indices.mapping.put.TransportPutMappingAction;
|
||||||
|
import org.elasticsearch.action.admin.indices.open.TransportOpenIndexAction;
|
||||||
import org.elasticsearch.action.admin.indices.optimize.TransportOptimizeAction;
|
import org.elasticsearch.action.admin.indices.optimize.TransportOptimizeAction;
|
||||||
import org.elasticsearch.action.admin.indices.refresh.TransportRefreshAction;
|
import org.elasticsearch.action.admin.indices.refresh.TransportRefreshAction;
|
||||||
import org.elasticsearch.action.admin.indices.settings.TransportUpdateSettingsAction;
|
import org.elasticsearch.action.admin.indices.settings.TransportUpdateSettingsAction;
|
||||||
|
@ -80,6 +82,8 @@ public class TransportActionModule extends AbstractModule {
|
||||||
bind(TransportIndicesStatusAction.class).asEagerSingleton();
|
bind(TransportIndicesStatusAction.class).asEagerSingleton();
|
||||||
bind(TransportCreateIndexAction.class).asEagerSingleton();
|
bind(TransportCreateIndexAction.class).asEagerSingleton();
|
||||||
bind(TransportDeleteIndexAction.class).asEagerSingleton();
|
bind(TransportDeleteIndexAction.class).asEagerSingleton();
|
||||||
|
bind(TransportOpenIndexAction.class).asEagerSingleton();
|
||||||
|
bind(TransportCloseIndexAction.class).asEagerSingleton();
|
||||||
bind(TransportPutMappingAction.class).asEagerSingleton();
|
bind(TransportPutMappingAction.class).asEagerSingleton();
|
||||||
bind(TransportDeleteMappingAction.class).asEagerSingleton();
|
bind(TransportDeleteMappingAction.class).asEagerSingleton();
|
||||||
bind(TransportIndicesAliasesAction.class).asEagerSingleton();
|
bind(TransportIndicesAliasesAction.class).asEagerSingleton();
|
||||||
|
|
|
@ -49,6 +49,8 @@ public class TransportActions {
|
||||||
public static class Indices {
|
public static class Indices {
|
||||||
public static final String CREATE = "indices/createIndex";
|
public static final String CREATE = "indices/createIndex";
|
||||||
public static final String DELETE = "indices/deleteIndex";
|
public static final String DELETE = "indices/deleteIndex";
|
||||||
|
public static final String OPEN = "indices/openIndex";
|
||||||
|
public static final String CLOSE = "indices/closeIndex";
|
||||||
public static final String FLUSH = "indices/flush";
|
public static final String FLUSH = "indices/flush";
|
||||||
public static final String REFRESH = "indices/refresh";
|
public static final String REFRESH = "indices/refresh";
|
||||||
public static final String OPTIMIZE = "indices/optimize";
|
public static final String OPTIMIZE = "indices/optimize";
|
||||||
|
|
|
@ -0,0 +1,105 @@
|
||||||
|
/*
|
||||||
|
* 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.action.admin.indices.close;
|
||||||
|
|
||||||
|
import org.elasticsearch.action.ActionRequestValidationException;
|
||||||
|
import org.elasticsearch.action.support.master.MasterNodeOperationRequest;
|
||||||
|
import org.elasticsearch.common.io.stream.StreamInput;
|
||||||
|
import org.elasticsearch.common.io.stream.StreamOutput;
|
||||||
|
import org.elasticsearch.common.unit.TimeValue;
|
||||||
|
|
||||||
|
import java.io.IOException;
|
||||||
|
|
||||||
|
import static org.elasticsearch.action.Actions.*;
|
||||||
|
import static org.elasticsearch.common.unit.TimeValue.*;
|
||||||
|
|
||||||
|
/**
|
||||||
|
* A request to close an index.
|
||||||
|
*
|
||||||
|
* @author kimchy (shay.banon)
|
||||||
|
*/
|
||||||
|
public class CloseIndexRequest extends MasterNodeOperationRequest {
|
||||||
|
|
||||||
|
private String index;
|
||||||
|
|
||||||
|
private TimeValue timeout = timeValueSeconds(10);
|
||||||
|
|
||||||
|
CloseIndexRequest() {
|
||||||
|
}
|
||||||
|
|
||||||
|
/**
|
||||||
|
* Constructs a new delete index request for the specified index.
|
||||||
|
*/
|
||||||
|
public CloseIndexRequest(String index) {
|
||||||
|
this.index = index;
|
||||||
|
}
|
||||||
|
|
||||||
|
@Override public ActionRequestValidationException validate() {
|
||||||
|
ActionRequestValidationException validationException = null;
|
||||||
|
if (index == null) {
|
||||||
|
validationException = addValidationError("index is missing", validationException);
|
||||||
|
}
|
||||||
|
return validationException;
|
||||||
|
}
|
||||||
|
|
||||||
|
/**
|
||||||
|
* The index to delete.
|
||||||
|
*/
|
||||||
|
String index() {
|
||||||
|
return index;
|
||||||
|
}
|
||||||
|
|
||||||
|
/**
|
||||||
|
* Timeout to wait for the index deletion to be acknowledged by current cluster nodes. Defaults
|
||||||
|
* to <tt>10s</tt>.
|
||||||
|
*/
|
||||||
|
TimeValue timeout() {
|
||||||
|
return timeout;
|
||||||
|
}
|
||||||
|
|
||||||
|
/**
|
||||||
|
* Timeout to wait for the index deletion to be acknowledged by current cluster nodes. Defaults
|
||||||
|
* to <tt>10s</tt>.
|
||||||
|
*/
|
||||||
|
public CloseIndexRequest timeout(TimeValue timeout) {
|
||||||
|
this.timeout = timeout;
|
||||||
|
return this;
|
||||||
|
}
|
||||||
|
|
||||||
|
/**
|
||||||
|
* Timeout to wait for the index deletion to be acknowledged by current cluster nodes. Defaults
|
||||||
|
* to <tt>10s</tt>.
|
||||||
|
*/
|
||||||
|
public CloseIndexRequest timeout(String timeout) {
|
||||||
|
return timeout(TimeValue.parseTimeValue(timeout, null));
|
||||||
|
}
|
||||||
|
|
||||||
|
@Override public void readFrom(StreamInput in) throws IOException {
|
||||||
|
super.readFrom(in);
|
||||||
|
index = in.readUTF();
|
||||||
|
timeout = readTimeValue(in);
|
||||||
|
}
|
||||||
|
|
||||||
|
@Override public void writeTo(StreamOutput out) throws IOException {
|
||||||
|
super.writeTo(out);
|
||||||
|
out.writeUTF(index);
|
||||||
|
timeout.writeTo(out);
|
||||||
|
}
|
||||||
|
}
|
|
@ -0,0 +1,60 @@
|
||||||
|
/*
|
||||||
|
* 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.action.admin.indices.close;
|
||||||
|
|
||||||
|
import org.elasticsearch.action.ActionResponse;
|
||||||
|
import org.elasticsearch.common.io.stream.StreamInput;
|
||||||
|
import org.elasticsearch.common.io.stream.StreamOutput;
|
||||||
|
import org.elasticsearch.common.io.stream.Streamable;
|
||||||
|
|
||||||
|
import java.io.IOException;
|
||||||
|
|
||||||
|
/**
|
||||||
|
* A response for a close index action.
|
||||||
|
*
|
||||||
|
* @author kimchy (shay.banon)
|
||||||
|
*/
|
||||||
|
public class CloseIndexResponse implements ActionResponse, Streamable {
|
||||||
|
|
||||||
|
private boolean acknowledged;
|
||||||
|
|
||||||
|
CloseIndexResponse() {
|
||||||
|
}
|
||||||
|
|
||||||
|
CloseIndexResponse(boolean acknowledged) {
|
||||||
|
this.acknowledged = acknowledged;
|
||||||
|
}
|
||||||
|
|
||||||
|
public boolean acknowledged() {
|
||||||
|
return acknowledged;
|
||||||
|
}
|
||||||
|
|
||||||
|
public boolean getAcknowledged() {
|
||||||
|
return acknowledged();
|
||||||
|
}
|
||||||
|
|
||||||
|
@Override public void readFrom(StreamInput in) throws IOException {
|
||||||
|
acknowledged = in.readBoolean();
|
||||||
|
}
|
||||||
|
|
||||||
|
@Override public void writeTo(StreamOutput out) throws IOException {
|
||||||
|
out.writeBoolean(acknowledged);
|
||||||
|
}
|
||||||
|
}
|
|
@ -0,0 +1,100 @@
|
||||||
|
/*
|
||||||
|
* 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.action.admin.indices.close;
|
||||||
|
|
||||||
|
import org.elasticsearch.ElasticSearchException;
|
||||||
|
import org.elasticsearch.action.TransportActions;
|
||||||
|
import org.elasticsearch.action.support.master.TransportMasterNodeOperationAction;
|
||||||
|
import org.elasticsearch.cluster.ClusterService;
|
||||||
|
import org.elasticsearch.cluster.ClusterState;
|
||||||
|
import org.elasticsearch.cluster.block.ClusterBlockLevel;
|
||||||
|
import org.elasticsearch.cluster.metadata.MetaDataStateIndexService;
|
||||||
|
import org.elasticsearch.common.inject.Inject;
|
||||||
|
import org.elasticsearch.common.settings.Settings;
|
||||||
|
import org.elasticsearch.threadpool.ThreadPool;
|
||||||
|
import org.elasticsearch.transport.TransportService;
|
||||||
|
|
||||||
|
import java.util.concurrent.CountDownLatch;
|
||||||
|
import java.util.concurrent.atomic.AtomicReference;
|
||||||
|
|
||||||
|
/**
|
||||||
|
* Delete index action.
|
||||||
|
*
|
||||||
|
* @author kimchy (shay.banon)
|
||||||
|
*/
|
||||||
|
public class TransportCloseIndexAction extends TransportMasterNodeOperationAction<CloseIndexRequest, CloseIndexResponse> {
|
||||||
|
|
||||||
|
private final MetaDataStateIndexService stateIndexService;
|
||||||
|
|
||||||
|
@Inject public TransportCloseIndexAction(Settings settings, TransportService transportService, ClusterService clusterService,
|
||||||
|
ThreadPool threadPool, MetaDataStateIndexService stateIndexService) {
|
||||||
|
super(settings, transportService, clusterService, threadPool);
|
||||||
|
this.stateIndexService = stateIndexService;
|
||||||
|
}
|
||||||
|
|
||||||
|
@Override protected String transportAction() {
|
||||||
|
return TransportActions.Admin.Indices.CLOSE;
|
||||||
|
}
|
||||||
|
|
||||||
|
@Override protected CloseIndexRequest newRequest() {
|
||||||
|
return new CloseIndexRequest();
|
||||||
|
}
|
||||||
|
|
||||||
|
@Override protected CloseIndexResponse newResponse() {
|
||||||
|
return new CloseIndexResponse();
|
||||||
|
}
|
||||||
|
|
||||||
|
@Override protected void checkBlock(CloseIndexRequest request, ClusterState state) {
|
||||||
|
state.blocks().indexBlockedRaiseException(ClusterBlockLevel.METADATA, request.index());
|
||||||
|
}
|
||||||
|
|
||||||
|
@Override protected CloseIndexResponse masterOperation(CloseIndexRequest request, ClusterState state) throws ElasticSearchException {
|
||||||
|
final AtomicReference<CloseIndexResponse> responseRef = new AtomicReference<CloseIndexResponse>();
|
||||||
|
final AtomicReference<Throwable> failureRef = new AtomicReference<Throwable>();
|
||||||
|
final CountDownLatch latch = new CountDownLatch(1);
|
||||||
|
stateIndexService.closeIndex(new MetaDataStateIndexService.Request(request.index()).timeout(request.timeout()), new MetaDataStateIndexService.Listener() {
|
||||||
|
@Override public void onResponse(MetaDataStateIndexService.Response response) {
|
||||||
|
responseRef.set(new CloseIndexResponse(response.acknowledged()));
|
||||||
|
latch.countDown();
|
||||||
|
}
|
||||||
|
|
||||||
|
@Override public void onFailure(Throwable t) {
|
||||||
|
failureRef.set(t);
|
||||||
|
latch.countDown();
|
||||||
|
}
|
||||||
|
});
|
||||||
|
|
||||||
|
try {
|
||||||
|
latch.await();
|
||||||
|
} catch (InterruptedException e) {
|
||||||
|
failureRef.set(e);
|
||||||
|
}
|
||||||
|
|
||||||
|
if (failureRef.get() != null) {
|
||||||
|
if (failureRef.get() instanceof ElasticSearchException) {
|
||||||
|
throw (ElasticSearchException) failureRef.get();
|
||||||
|
} else {
|
||||||
|
throw new ElasticSearchException(failureRef.get().getMessage(), failureRef.get());
|
||||||
|
}
|
||||||
|
}
|
||||||
|
|
||||||
|
return responseRef.get();
|
||||||
|
}
|
||||||
|
}
|
|
@ -0,0 +1,105 @@
|
||||||
|
/*
|
||||||
|
* 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.action.admin.indices.open;
|
||||||
|
|
||||||
|
import org.elasticsearch.action.ActionRequestValidationException;
|
||||||
|
import org.elasticsearch.action.support.master.MasterNodeOperationRequest;
|
||||||
|
import org.elasticsearch.common.io.stream.StreamInput;
|
||||||
|
import org.elasticsearch.common.io.stream.StreamOutput;
|
||||||
|
import org.elasticsearch.common.unit.TimeValue;
|
||||||
|
|
||||||
|
import java.io.IOException;
|
||||||
|
|
||||||
|
import static org.elasticsearch.action.Actions.*;
|
||||||
|
import static org.elasticsearch.common.unit.TimeValue.*;
|
||||||
|
|
||||||
|
/**
|
||||||
|
* A request to open an index.
|
||||||
|
*
|
||||||
|
* @author kimchy (shay.banon)
|
||||||
|
*/
|
||||||
|
public class OpenIndexRequest extends MasterNodeOperationRequest {
|
||||||
|
|
||||||
|
private String index;
|
||||||
|
|
||||||
|
private TimeValue timeout = timeValueSeconds(10);
|
||||||
|
|
||||||
|
OpenIndexRequest() {
|
||||||
|
}
|
||||||
|
|
||||||
|
/**
|
||||||
|
* Constructs a new delete index request for the specified index.
|
||||||
|
*/
|
||||||
|
public OpenIndexRequest(String index) {
|
||||||
|
this.index = index;
|
||||||
|
}
|
||||||
|
|
||||||
|
@Override public ActionRequestValidationException validate() {
|
||||||
|
ActionRequestValidationException validationException = null;
|
||||||
|
if (index == null) {
|
||||||
|
validationException = addValidationError("index is missing", validationException);
|
||||||
|
}
|
||||||
|
return validationException;
|
||||||
|
}
|
||||||
|
|
||||||
|
/**
|
||||||
|
* The index to delete.
|
||||||
|
*/
|
||||||
|
String index() {
|
||||||
|
return index;
|
||||||
|
}
|
||||||
|
|
||||||
|
/**
|
||||||
|
* Timeout to wait for the index deletion to be acknowledged by current cluster nodes. Defaults
|
||||||
|
* to <tt>10s</tt>.
|
||||||
|
*/
|
||||||
|
TimeValue timeout() {
|
||||||
|
return timeout;
|
||||||
|
}
|
||||||
|
|
||||||
|
/**
|
||||||
|
* Timeout to wait for the index deletion to be acknowledged by current cluster nodes. Defaults
|
||||||
|
* to <tt>10s</tt>.
|
||||||
|
*/
|
||||||
|
public OpenIndexRequest timeout(TimeValue timeout) {
|
||||||
|
this.timeout = timeout;
|
||||||
|
return this;
|
||||||
|
}
|
||||||
|
|
||||||
|
/**
|
||||||
|
* Timeout to wait for the index deletion to be acknowledged by current cluster nodes. Defaults
|
||||||
|
* to <tt>10s</tt>.
|
||||||
|
*/
|
||||||
|
public OpenIndexRequest timeout(String timeout) {
|
||||||
|
return timeout(TimeValue.parseTimeValue(timeout, null));
|
||||||
|
}
|
||||||
|
|
||||||
|
@Override public void readFrom(StreamInput in) throws IOException {
|
||||||
|
super.readFrom(in);
|
||||||
|
index = in.readUTF();
|
||||||
|
timeout = readTimeValue(in);
|
||||||
|
}
|
||||||
|
|
||||||
|
@Override public void writeTo(StreamOutput out) throws IOException {
|
||||||
|
super.writeTo(out);
|
||||||
|
out.writeUTF(index);
|
||||||
|
timeout.writeTo(out);
|
||||||
|
}
|
||||||
|
}
|
|
@ -0,0 +1,60 @@
|
||||||
|
/*
|
||||||
|
* 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.action.admin.indices.open;
|
||||||
|
|
||||||
|
import org.elasticsearch.action.ActionResponse;
|
||||||
|
import org.elasticsearch.common.io.stream.StreamInput;
|
||||||
|
import org.elasticsearch.common.io.stream.StreamOutput;
|
||||||
|
import org.elasticsearch.common.io.stream.Streamable;
|
||||||
|
|
||||||
|
import java.io.IOException;
|
||||||
|
|
||||||
|
/**
|
||||||
|
* A response for a open index action.
|
||||||
|
*
|
||||||
|
* @author kimchy (shay.banon)
|
||||||
|
*/
|
||||||
|
public class OpenIndexResponse implements ActionResponse, Streamable {
|
||||||
|
|
||||||
|
private boolean acknowledged;
|
||||||
|
|
||||||
|
OpenIndexResponse() {
|
||||||
|
}
|
||||||
|
|
||||||
|
OpenIndexResponse(boolean acknowledged) {
|
||||||
|
this.acknowledged = acknowledged;
|
||||||
|
}
|
||||||
|
|
||||||
|
public boolean acknowledged() {
|
||||||
|
return acknowledged;
|
||||||
|
}
|
||||||
|
|
||||||
|
public boolean getAcknowledged() {
|
||||||
|
return acknowledged();
|
||||||
|
}
|
||||||
|
|
||||||
|
@Override public void readFrom(StreamInput in) throws IOException {
|
||||||
|
acknowledged = in.readBoolean();
|
||||||
|
}
|
||||||
|
|
||||||
|
@Override public void writeTo(StreamOutput out) throws IOException {
|
||||||
|
out.writeBoolean(acknowledged);
|
||||||
|
}
|
||||||
|
}
|
|
@ -0,0 +1,100 @@
|
||||||
|
/*
|
||||||
|
* 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.action.admin.indices.open;
|
||||||
|
|
||||||
|
import org.elasticsearch.ElasticSearchException;
|
||||||
|
import org.elasticsearch.action.TransportActions;
|
||||||
|
import org.elasticsearch.action.support.master.TransportMasterNodeOperationAction;
|
||||||
|
import org.elasticsearch.cluster.ClusterService;
|
||||||
|
import org.elasticsearch.cluster.ClusterState;
|
||||||
|
import org.elasticsearch.cluster.block.ClusterBlockLevel;
|
||||||
|
import org.elasticsearch.cluster.metadata.MetaDataStateIndexService;
|
||||||
|
import org.elasticsearch.common.inject.Inject;
|
||||||
|
import org.elasticsearch.common.settings.Settings;
|
||||||
|
import org.elasticsearch.threadpool.ThreadPool;
|
||||||
|
import org.elasticsearch.transport.TransportService;
|
||||||
|
|
||||||
|
import java.util.concurrent.CountDownLatch;
|
||||||
|
import java.util.concurrent.atomic.AtomicReference;
|
||||||
|
|
||||||
|
/**
|
||||||
|
* Delete index action.
|
||||||
|
*
|
||||||
|
* @author kimchy (shay.banon)
|
||||||
|
*/
|
||||||
|
public class TransportOpenIndexAction extends TransportMasterNodeOperationAction<OpenIndexRequest, OpenIndexResponse> {
|
||||||
|
|
||||||
|
private final MetaDataStateIndexService stateIndexService;
|
||||||
|
|
||||||
|
@Inject public TransportOpenIndexAction(Settings settings, TransportService transportService, ClusterService clusterService,
|
||||||
|
ThreadPool threadPool, MetaDataStateIndexService stateIndexService) {
|
||||||
|
super(settings, transportService, clusterService, threadPool);
|
||||||
|
this.stateIndexService = stateIndexService;
|
||||||
|
}
|
||||||
|
|
||||||
|
@Override protected String transportAction() {
|
||||||
|
return TransportActions.Admin.Indices.OPEN;
|
||||||
|
}
|
||||||
|
|
||||||
|
@Override protected OpenIndexRequest newRequest() {
|
||||||
|
return new OpenIndexRequest();
|
||||||
|
}
|
||||||
|
|
||||||
|
@Override protected OpenIndexResponse newResponse() {
|
||||||
|
return new OpenIndexResponse();
|
||||||
|
}
|
||||||
|
|
||||||
|
@Override protected void checkBlock(OpenIndexRequest request, ClusterState state) {
|
||||||
|
state.blocks().indexBlockedRaiseException(ClusterBlockLevel.METADATA, request.index());
|
||||||
|
}
|
||||||
|
|
||||||
|
@Override protected OpenIndexResponse masterOperation(OpenIndexRequest request, ClusterState state) throws ElasticSearchException {
|
||||||
|
final AtomicReference<OpenIndexResponse> responseRef = new AtomicReference<OpenIndexResponse>();
|
||||||
|
final AtomicReference<Throwable> failureRef = new AtomicReference<Throwable>();
|
||||||
|
final CountDownLatch latch = new CountDownLatch(1);
|
||||||
|
stateIndexService.openIndex(new MetaDataStateIndexService.Request(request.index()).timeout(request.timeout()), new MetaDataStateIndexService.Listener() {
|
||||||
|
@Override public void onResponse(MetaDataStateIndexService.Response response) {
|
||||||
|
responseRef.set(new OpenIndexResponse(response.acknowledged()));
|
||||||
|
latch.countDown();
|
||||||
|
}
|
||||||
|
|
||||||
|
@Override public void onFailure(Throwable t) {
|
||||||
|
failureRef.set(t);
|
||||||
|
latch.countDown();
|
||||||
|
}
|
||||||
|
});
|
||||||
|
|
||||||
|
try {
|
||||||
|
latch.await();
|
||||||
|
} catch (InterruptedException e) {
|
||||||
|
failureRef.set(e);
|
||||||
|
}
|
||||||
|
|
||||||
|
if (failureRef.get() != null) {
|
||||||
|
if (failureRef.get() instanceof ElasticSearchException) {
|
||||||
|
throw (ElasticSearchException) failureRef.get();
|
||||||
|
} else {
|
||||||
|
throw new ElasticSearchException(failureRef.get().getMessage(), failureRef.get());
|
||||||
|
}
|
||||||
|
}
|
||||||
|
|
||||||
|
return responseRef.get();
|
||||||
|
}
|
||||||
|
}
|
|
@ -25,6 +25,8 @@ import org.elasticsearch.action.admin.indices.alias.IndicesAliasesRequest;
|
||||||
import org.elasticsearch.action.admin.indices.alias.IndicesAliasesResponse;
|
import org.elasticsearch.action.admin.indices.alias.IndicesAliasesResponse;
|
||||||
import org.elasticsearch.action.admin.indices.cache.clear.ClearIndicesCacheRequest;
|
import org.elasticsearch.action.admin.indices.cache.clear.ClearIndicesCacheRequest;
|
||||||
import org.elasticsearch.action.admin.indices.cache.clear.ClearIndicesCacheResponse;
|
import org.elasticsearch.action.admin.indices.cache.clear.ClearIndicesCacheResponse;
|
||||||
|
import org.elasticsearch.action.admin.indices.close.CloseIndexRequest;
|
||||||
|
import org.elasticsearch.action.admin.indices.close.CloseIndexResponse;
|
||||||
import org.elasticsearch.action.admin.indices.create.CreateIndexRequest;
|
import org.elasticsearch.action.admin.indices.create.CreateIndexRequest;
|
||||||
import org.elasticsearch.action.admin.indices.create.CreateIndexResponse;
|
import org.elasticsearch.action.admin.indices.create.CreateIndexResponse;
|
||||||
import org.elasticsearch.action.admin.indices.delete.DeleteIndexRequest;
|
import org.elasticsearch.action.admin.indices.delete.DeleteIndexRequest;
|
||||||
|
@ -37,6 +39,8 @@ import org.elasticsearch.action.admin.indices.mapping.delete.DeleteMappingReques
|
||||||
import org.elasticsearch.action.admin.indices.mapping.delete.DeleteMappingResponse;
|
import org.elasticsearch.action.admin.indices.mapping.delete.DeleteMappingResponse;
|
||||||
import org.elasticsearch.action.admin.indices.mapping.put.PutMappingRequest;
|
import org.elasticsearch.action.admin.indices.mapping.put.PutMappingRequest;
|
||||||
import org.elasticsearch.action.admin.indices.mapping.put.PutMappingResponse;
|
import org.elasticsearch.action.admin.indices.mapping.put.PutMappingResponse;
|
||||||
|
import org.elasticsearch.action.admin.indices.open.OpenIndexRequest;
|
||||||
|
import org.elasticsearch.action.admin.indices.open.OpenIndexResponse;
|
||||||
import org.elasticsearch.action.admin.indices.optimize.OptimizeRequest;
|
import org.elasticsearch.action.admin.indices.optimize.OptimizeRequest;
|
||||||
import org.elasticsearch.action.admin.indices.optimize.OptimizeResponse;
|
import org.elasticsearch.action.admin.indices.optimize.OptimizeResponse;
|
||||||
import org.elasticsearch.action.admin.indices.refresh.RefreshRequest;
|
import org.elasticsearch.action.admin.indices.refresh.RefreshRequest;
|
||||||
|
@ -47,12 +51,14 @@ import org.elasticsearch.action.admin.indices.status.IndicesStatusRequest;
|
||||||
import org.elasticsearch.action.admin.indices.status.IndicesStatusResponse;
|
import org.elasticsearch.action.admin.indices.status.IndicesStatusResponse;
|
||||||
import org.elasticsearch.client.action.admin.indices.alias.IndicesAliasesRequestBuilder;
|
import org.elasticsearch.client.action.admin.indices.alias.IndicesAliasesRequestBuilder;
|
||||||
import org.elasticsearch.client.action.admin.indices.cache.clear.ClearIndicesCacheRequestBuilder;
|
import org.elasticsearch.client.action.admin.indices.cache.clear.ClearIndicesCacheRequestBuilder;
|
||||||
|
import org.elasticsearch.client.action.admin.indices.close.CloseIndexRequestBuilder;
|
||||||
import org.elasticsearch.client.action.admin.indices.create.CreateIndexRequestBuilder;
|
import org.elasticsearch.client.action.admin.indices.create.CreateIndexRequestBuilder;
|
||||||
import org.elasticsearch.client.action.admin.indices.delete.DeleteIndexRequestBuilder;
|
import org.elasticsearch.client.action.admin.indices.delete.DeleteIndexRequestBuilder;
|
||||||
import org.elasticsearch.client.action.admin.indices.flush.FlushRequestBuilder;
|
import org.elasticsearch.client.action.admin.indices.flush.FlushRequestBuilder;
|
||||||
import org.elasticsearch.client.action.admin.indices.gateway.snapshot.GatewaySnapshotRequestBuilder;
|
import org.elasticsearch.client.action.admin.indices.gateway.snapshot.GatewaySnapshotRequestBuilder;
|
||||||
import org.elasticsearch.client.action.admin.indices.mapping.delete.DeleteMappingRequestBuilder;
|
import org.elasticsearch.client.action.admin.indices.mapping.delete.DeleteMappingRequestBuilder;
|
||||||
import org.elasticsearch.client.action.admin.indices.mapping.put.PutMappingRequestBuilder;
|
import org.elasticsearch.client.action.admin.indices.mapping.put.PutMappingRequestBuilder;
|
||||||
|
import org.elasticsearch.client.action.admin.indices.open.OpenIndexRequestBuilder;
|
||||||
import org.elasticsearch.client.action.admin.indices.optimize.OptimizeRequestBuilder;
|
import org.elasticsearch.client.action.admin.indices.optimize.OptimizeRequestBuilder;
|
||||||
import org.elasticsearch.client.action.admin.indices.refresh.RefreshRequestBuilder;
|
import org.elasticsearch.client.action.admin.indices.refresh.RefreshRequestBuilder;
|
||||||
import org.elasticsearch.client.action.admin.indices.settings.UpdateSettingsRequestBuilder;
|
import org.elasticsearch.client.action.admin.indices.settings.UpdateSettingsRequestBuilder;
|
||||||
|
@ -139,6 +145,56 @@ public interface IndicesAdminClient {
|
||||||
*/
|
*/
|
||||||
DeleteIndexRequestBuilder prepareDelete(String index);
|
DeleteIndexRequestBuilder prepareDelete(String index);
|
||||||
|
|
||||||
|
/**
|
||||||
|
* Closes an index based on the index name.
|
||||||
|
*
|
||||||
|
* @param request The close index request
|
||||||
|
* @return The result future
|
||||||
|
* @see org.elasticsearch.client.Requests#closeIndexRequest(String)
|
||||||
|
*/
|
||||||
|
ActionFuture<CloseIndexResponse> close(CloseIndexRequest request);
|
||||||
|
|
||||||
|
/**
|
||||||
|
* Closes an index based on the index name.
|
||||||
|
*
|
||||||
|
* @param request The close index request
|
||||||
|
* @param listener A listener to be notified with a result
|
||||||
|
* @see org.elasticsearch.client.Requests#closeIndexRequest(String)
|
||||||
|
*/
|
||||||
|
void close(CloseIndexRequest request, ActionListener<CloseIndexResponse> listener);
|
||||||
|
|
||||||
|
/**
|
||||||
|
* Closes an index based on the index name.
|
||||||
|
*
|
||||||
|
* @param index The index name to close
|
||||||
|
*/
|
||||||
|
CloseIndexRequestBuilder prepareClose(String index);
|
||||||
|
|
||||||
|
/**
|
||||||
|
* OPen an index based on the index name.
|
||||||
|
*
|
||||||
|
* @param request The close index request
|
||||||
|
* @return The result future
|
||||||
|
* @see org.elasticsearch.client.Requests#openIndexRequest(String)
|
||||||
|
*/
|
||||||
|
ActionFuture<OpenIndexResponse> open(OpenIndexRequest request);
|
||||||
|
|
||||||
|
/**
|
||||||
|
* Open an index based on the index name.
|
||||||
|
*
|
||||||
|
* @param request The close index request
|
||||||
|
* @param listener A listener to be notified with a result
|
||||||
|
* @see org.elasticsearch.client.Requests#openIndexRequest(String)
|
||||||
|
*/
|
||||||
|
void open(OpenIndexRequest request, ActionListener<OpenIndexResponse> listener);
|
||||||
|
|
||||||
|
/**
|
||||||
|
* Opens an index based on the index name.
|
||||||
|
*
|
||||||
|
* @param index The index name to close
|
||||||
|
*/
|
||||||
|
OpenIndexRequestBuilder prepareOpen(String index);
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* Explicitly refresh one or more indices (making the content indexed since the last refresh searchable).
|
* Explicitly refresh one or more indices (making the content indexed since the last refresh searchable).
|
||||||
*
|
*
|
||||||
|
|
|
@ -30,12 +30,14 @@ import org.elasticsearch.action.admin.cluster.ping.single.SinglePingRequest;
|
||||||
import org.elasticsearch.action.admin.cluster.state.ClusterStateRequest;
|
import org.elasticsearch.action.admin.cluster.state.ClusterStateRequest;
|
||||||
import org.elasticsearch.action.admin.indices.alias.IndicesAliasesRequest;
|
import org.elasticsearch.action.admin.indices.alias.IndicesAliasesRequest;
|
||||||
import org.elasticsearch.action.admin.indices.cache.clear.ClearIndicesCacheRequest;
|
import org.elasticsearch.action.admin.indices.cache.clear.ClearIndicesCacheRequest;
|
||||||
|
import org.elasticsearch.action.admin.indices.close.CloseIndexRequest;
|
||||||
import org.elasticsearch.action.admin.indices.create.CreateIndexRequest;
|
import org.elasticsearch.action.admin.indices.create.CreateIndexRequest;
|
||||||
import org.elasticsearch.action.admin.indices.delete.DeleteIndexRequest;
|
import org.elasticsearch.action.admin.indices.delete.DeleteIndexRequest;
|
||||||
import org.elasticsearch.action.admin.indices.flush.FlushRequest;
|
import org.elasticsearch.action.admin.indices.flush.FlushRequest;
|
||||||
import org.elasticsearch.action.admin.indices.gateway.snapshot.GatewaySnapshotRequest;
|
import org.elasticsearch.action.admin.indices.gateway.snapshot.GatewaySnapshotRequest;
|
||||||
import org.elasticsearch.action.admin.indices.mapping.delete.DeleteMappingRequest;
|
import org.elasticsearch.action.admin.indices.mapping.delete.DeleteMappingRequest;
|
||||||
import org.elasticsearch.action.admin.indices.mapping.put.PutMappingRequest;
|
import org.elasticsearch.action.admin.indices.mapping.put.PutMappingRequest;
|
||||||
|
import org.elasticsearch.action.admin.indices.open.OpenIndexRequest;
|
||||||
import org.elasticsearch.action.admin.indices.optimize.OptimizeRequest;
|
import org.elasticsearch.action.admin.indices.optimize.OptimizeRequest;
|
||||||
import org.elasticsearch.action.admin.indices.refresh.RefreshRequest;
|
import org.elasticsearch.action.admin.indices.refresh.RefreshRequest;
|
||||||
import org.elasticsearch.action.admin.indices.settings.UpdateSettingsRequest;
|
import org.elasticsearch.action.admin.indices.settings.UpdateSettingsRequest;
|
||||||
|
@ -207,6 +209,28 @@ public class Requests {
|
||||||
return new DeleteIndexRequest(index);
|
return new DeleteIndexRequest(index);
|
||||||
}
|
}
|
||||||
|
|
||||||
|
/**
|
||||||
|
* Creates a close index request.
|
||||||
|
*
|
||||||
|
* @param index The index to close
|
||||||
|
* @return The delete index request
|
||||||
|
* @see org.elasticsearch.client.IndicesAdminClient#close(org.elasticsearch.action.admin.indices.close.CloseIndexRequest)
|
||||||
|
*/
|
||||||
|
public static CloseIndexRequest closeIndexRequest(String index) {
|
||||||
|
return new CloseIndexRequest(index);
|
||||||
|
}
|
||||||
|
|
||||||
|
/**
|
||||||
|
* Creates an open index request.
|
||||||
|
*
|
||||||
|
* @param index The index to open
|
||||||
|
* @return The delete index request
|
||||||
|
* @see org.elasticsearch.client.IndicesAdminClient#open(org.elasticsearch.action.admin.indices.open.OpenIndexRequest)
|
||||||
|
*/
|
||||||
|
public static OpenIndexRequest openIndexRequest(String index) {
|
||||||
|
return new OpenIndexRequest(index);
|
||||||
|
}
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* Create a create mapping request against one or more indices.
|
* Create a create mapping request against one or more indices.
|
||||||
*
|
*
|
||||||
|
@ -223,7 +247,7 @@ public class Requests {
|
||||||
*
|
*
|
||||||
* @param indices The indices the mapping will be deleted from. Use <tt>null</tt> or <tt>_all</tt> to execute against all indices
|
* @param indices The indices the mapping will be deleted from. Use <tt>null</tt> or <tt>_all</tt> to execute against all indices
|
||||||
* @return The create mapping request
|
* @return The create mapping request
|
||||||
* @see org.elasticsearch.client.IndicesAdminClient#deleteMapping(org.elasticsearch.action.admin.indices.mapping.put.DeleteMappingRequest)
|
* @see org.elasticsearch.client.IndicesAdminClient#deleteMapping(org.elasticsearch.action.admin.indices.mapping.delete.DeleteMappingRequest)
|
||||||
*/
|
*/
|
||||||
public static DeleteMappingRequest deleteMappingRequest(String... indices) {
|
public static DeleteMappingRequest deleteMappingRequest(String... indices) {
|
||||||
return new DeleteMappingRequest(indices);
|
return new DeleteMappingRequest(indices);
|
||||||
|
|
|
@ -0,0 +1,75 @@
|
||||||
|
/*
|
||||||
|
* 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.client.action.admin.indices.close;
|
||||||
|
|
||||||
|
import org.elasticsearch.action.ActionListener;
|
||||||
|
import org.elasticsearch.action.admin.indices.close.CloseIndexRequest;
|
||||||
|
import org.elasticsearch.action.admin.indices.close.CloseIndexResponse;
|
||||||
|
import org.elasticsearch.client.IndicesAdminClient;
|
||||||
|
import org.elasticsearch.client.action.admin.indices.support.BaseIndicesRequestBuilder;
|
||||||
|
import org.elasticsearch.common.unit.TimeValue;
|
||||||
|
|
||||||
|
/**
|
||||||
|
* @author kimchy (shay.banon)
|
||||||
|
*/
|
||||||
|
public class CloseIndexRequestBuilder extends BaseIndicesRequestBuilder<CloseIndexRequest, CloseIndexResponse> {
|
||||||
|
|
||||||
|
public CloseIndexRequestBuilder(IndicesAdminClient indicesClient, String index) {
|
||||||
|
super(indicesClient, new CloseIndexRequest(index));
|
||||||
|
}
|
||||||
|
|
||||||
|
/**
|
||||||
|
* Timeout to wait for the operation to be acknowledged by current cluster nodes. Defaults
|
||||||
|
* to <tt>10s</tt>.
|
||||||
|
*/
|
||||||
|
public CloseIndexRequestBuilder setTimeout(TimeValue timeout) {
|
||||||
|
request.timeout(timeout);
|
||||||
|
return this;
|
||||||
|
}
|
||||||
|
|
||||||
|
/**
|
||||||
|
* Timeout to wait for the index deletion to be acknowledged by current cluster nodes. Defaults
|
||||||
|
* to <tt>10s</tt>.
|
||||||
|
*/
|
||||||
|
public CloseIndexRequestBuilder setTimeout(String timeout) {
|
||||||
|
request.timeout(timeout);
|
||||||
|
return this;
|
||||||
|
}
|
||||||
|
|
||||||
|
/**
|
||||||
|
* Sets the master node timeout in case the master has not yet been discovered.
|
||||||
|
*/
|
||||||
|
public CloseIndexRequestBuilder setMasterNodeTimeout(TimeValue timeout) {
|
||||||
|
request.masterNodeTimeout(timeout);
|
||||||
|
return this;
|
||||||
|
}
|
||||||
|
|
||||||
|
/**
|
||||||
|
* Sets the master node timeout in case the master has not yet been discovered.
|
||||||
|
*/
|
||||||
|
public CloseIndexRequestBuilder setMasterNodeTimeout(String timeout) {
|
||||||
|
request.masterNodeTimeout(timeout);
|
||||||
|
return this;
|
||||||
|
}
|
||||||
|
|
||||||
|
@Override protected void doExecute(ActionListener<CloseIndexResponse> listener) {
|
||||||
|
client.close(request, listener);
|
||||||
|
}
|
||||||
|
}
|
|
@ -0,0 +1,75 @@
|
||||||
|
/*
|
||||||
|
* 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.client.action.admin.indices.open;
|
||||||
|
|
||||||
|
import org.elasticsearch.action.ActionListener;
|
||||||
|
import org.elasticsearch.action.admin.indices.open.OpenIndexRequest;
|
||||||
|
import org.elasticsearch.action.admin.indices.open.OpenIndexResponse;
|
||||||
|
import org.elasticsearch.client.IndicesAdminClient;
|
||||||
|
import org.elasticsearch.client.action.admin.indices.support.BaseIndicesRequestBuilder;
|
||||||
|
import org.elasticsearch.common.unit.TimeValue;
|
||||||
|
|
||||||
|
/**
|
||||||
|
* @author kimchy (shay.banon)
|
||||||
|
*/
|
||||||
|
public class OpenIndexRequestBuilder extends BaseIndicesRequestBuilder<OpenIndexRequest, OpenIndexResponse> {
|
||||||
|
|
||||||
|
public OpenIndexRequestBuilder(IndicesAdminClient indicesClient, String index) {
|
||||||
|
super(indicesClient, new OpenIndexRequest(index));
|
||||||
|
}
|
||||||
|
|
||||||
|
/**
|
||||||
|
* Timeout to wait for the operation to be acknowledged by current cluster nodes. Defaults
|
||||||
|
* to <tt>10s</tt>.
|
||||||
|
*/
|
||||||
|
public OpenIndexRequestBuilder setTimeout(TimeValue timeout) {
|
||||||
|
request.timeout(timeout);
|
||||||
|
return this;
|
||||||
|
}
|
||||||
|
|
||||||
|
/**
|
||||||
|
* Timeout to wait for the index deletion to be acknowledged by current cluster nodes. Defaults
|
||||||
|
* to <tt>10s</tt>.
|
||||||
|
*/
|
||||||
|
public OpenIndexRequestBuilder setTimeout(String timeout) {
|
||||||
|
request.timeout(timeout);
|
||||||
|
return this;
|
||||||
|
}
|
||||||
|
|
||||||
|
/**
|
||||||
|
* Sets the master node timeout in case the master has not yet been discovered.
|
||||||
|
*/
|
||||||
|
public OpenIndexRequestBuilder setMasterNodeTimeout(TimeValue timeout) {
|
||||||
|
request.masterNodeTimeout(timeout);
|
||||||
|
return this;
|
||||||
|
}
|
||||||
|
|
||||||
|
/**
|
||||||
|
* Sets the master node timeout in case the master has not yet been discovered.
|
||||||
|
*/
|
||||||
|
public OpenIndexRequestBuilder setMasterNodeTimeout(String timeout) {
|
||||||
|
request.masterNodeTimeout(timeout);
|
||||||
|
return this;
|
||||||
|
}
|
||||||
|
|
||||||
|
@Override protected void doExecute(ActionListener<OpenIndexResponse> listener) {
|
||||||
|
client.open(request, listener);
|
||||||
|
}
|
||||||
|
}
|
|
@ -27,6 +27,9 @@ import org.elasticsearch.action.admin.indices.alias.TransportIndicesAliasesActio
|
||||||
import org.elasticsearch.action.admin.indices.cache.clear.ClearIndicesCacheRequest;
|
import org.elasticsearch.action.admin.indices.cache.clear.ClearIndicesCacheRequest;
|
||||||
import org.elasticsearch.action.admin.indices.cache.clear.ClearIndicesCacheResponse;
|
import org.elasticsearch.action.admin.indices.cache.clear.ClearIndicesCacheResponse;
|
||||||
import org.elasticsearch.action.admin.indices.cache.clear.TransportClearIndicesCacheAction;
|
import org.elasticsearch.action.admin.indices.cache.clear.TransportClearIndicesCacheAction;
|
||||||
|
import org.elasticsearch.action.admin.indices.close.CloseIndexRequest;
|
||||||
|
import org.elasticsearch.action.admin.indices.close.CloseIndexResponse;
|
||||||
|
import org.elasticsearch.action.admin.indices.close.TransportCloseIndexAction;
|
||||||
import org.elasticsearch.action.admin.indices.create.CreateIndexRequest;
|
import org.elasticsearch.action.admin.indices.create.CreateIndexRequest;
|
||||||
import org.elasticsearch.action.admin.indices.create.CreateIndexResponse;
|
import org.elasticsearch.action.admin.indices.create.CreateIndexResponse;
|
||||||
import org.elasticsearch.action.admin.indices.create.TransportCreateIndexAction;
|
import org.elasticsearch.action.admin.indices.create.TransportCreateIndexAction;
|
||||||
|
@ -45,6 +48,9 @@ import org.elasticsearch.action.admin.indices.mapping.delete.TransportDeleteMapp
|
||||||
import org.elasticsearch.action.admin.indices.mapping.put.PutMappingRequest;
|
import org.elasticsearch.action.admin.indices.mapping.put.PutMappingRequest;
|
||||||
import org.elasticsearch.action.admin.indices.mapping.put.PutMappingResponse;
|
import org.elasticsearch.action.admin.indices.mapping.put.PutMappingResponse;
|
||||||
import org.elasticsearch.action.admin.indices.mapping.put.TransportPutMappingAction;
|
import org.elasticsearch.action.admin.indices.mapping.put.TransportPutMappingAction;
|
||||||
|
import org.elasticsearch.action.admin.indices.open.OpenIndexRequest;
|
||||||
|
import org.elasticsearch.action.admin.indices.open.OpenIndexResponse;
|
||||||
|
import org.elasticsearch.action.admin.indices.open.TransportOpenIndexAction;
|
||||||
import org.elasticsearch.action.admin.indices.optimize.OptimizeRequest;
|
import org.elasticsearch.action.admin.indices.optimize.OptimizeRequest;
|
||||||
import org.elasticsearch.action.admin.indices.optimize.OptimizeResponse;
|
import org.elasticsearch.action.admin.indices.optimize.OptimizeResponse;
|
||||||
import org.elasticsearch.action.admin.indices.optimize.TransportOptimizeAction;
|
import org.elasticsearch.action.admin.indices.optimize.TransportOptimizeAction;
|
||||||
|
@ -76,6 +82,10 @@ public class NodeIndicesAdminClient extends AbstractIndicesAdminClient implement
|
||||||
|
|
||||||
private final TransportDeleteIndexAction deleteIndexAction;
|
private final TransportDeleteIndexAction deleteIndexAction;
|
||||||
|
|
||||||
|
private final TransportCloseIndexAction closeIndexAction;
|
||||||
|
|
||||||
|
private final TransportOpenIndexAction openIndexAction;
|
||||||
|
|
||||||
private final TransportRefreshAction refreshAction;
|
private final TransportRefreshAction refreshAction;
|
||||||
|
|
||||||
private final TransportFlushAction flushAction;
|
private final TransportFlushAction flushAction;
|
||||||
|
@ -96,6 +106,7 @@ public class NodeIndicesAdminClient extends AbstractIndicesAdminClient implement
|
||||||
|
|
||||||
@Inject public NodeIndicesAdminClient(Settings settings, ThreadPool threadPool, TransportIndicesStatusAction indicesStatusAction,
|
@Inject public NodeIndicesAdminClient(Settings settings, ThreadPool threadPool, TransportIndicesStatusAction indicesStatusAction,
|
||||||
TransportCreateIndexAction createIndexAction, TransportDeleteIndexAction deleteIndexAction,
|
TransportCreateIndexAction createIndexAction, TransportDeleteIndexAction deleteIndexAction,
|
||||||
|
TransportCloseIndexAction closeIndexAction, TransportOpenIndexAction openIndexAction,
|
||||||
TransportRefreshAction refreshAction, TransportFlushAction flushAction, TransportOptimizeAction optimizeAction,
|
TransportRefreshAction refreshAction, TransportFlushAction flushAction, TransportOptimizeAction optimizeAction,
|
||||||
TransportPutMappingAction putMappingAction, TransportDeleteMappingAction deleteMappingAction, TransportGatewaySnapshotAction gatewaySnapshotAction,
|
TransportPutMappingAction putMappingAction, TransportDeleteMappingAction deleteMappingAction, TransportGatewaySnapshotAction gatewaySnapshotAction,
|
||||||
TransportIndicesAliasesAction indicesAliasesAction, TransportClearIndicesCacheAction clearIndicesCacheAction,
|
TransportIndicesAliasesAction indicesAliasesAction, TransportClearIndicesCacheAction clearIndicesCacheAction,
|
||||||
|
@ -104,6 +115,8 @@ public class NodeIndicesAdminClient extends AbstractIndicesAdminClient implement
|
||||||
this.indicesStatusAction = indicesStatusAction;
|
this.indicesStatusAction = indicesStatusAction;
|
||||||
this.createIndexAction = createIndexAction;
|
this.createIndexAction = createIndexAction;
|
||||||
this.deleteIndexAction = deleteIndexAction;
|
this.deleteIndexAction = deleteIndexAction;
|
||||||
|
this.closeIndexAction = closeIndexAction;
|
||||||
|
this.openIndexAction = openIndexAction;
|
||||||
this.refreshAction = refreshAction;
|
this.refreshAction = refreshAction;
|
||||||
this.flushAction = flushAction;
|
this.flushAction = flushAction;
|
||||||
this.optimizeAction = optimizeAction;
|
this.optimizeAction = optimizeAction;
|
||||||
|
@ -143,6 +156,22 @@ public class NodeIndicesAdminClient extends AbstractIndicesAdminClient implement
|
||||||
deleteIndexAction.execute(request, listener);
|
deleteIndexAction.execute(request, listener);
|
||||||
}
|
}
|
||||||
|
|
||||||
|
@Override public ActionFuture<CloseIndexResponse> close(CloseIndexRequest request) {
|
||||||
|
return closeIndexAction.execute(request);
|
||||||
|
}
|
||||||
|
|
||||||
|
@Override public void close(CloseIndexRequest request, ActionListener<CloseIndexResponse> listener) {
|
||||||
|
closeIndexAction.execute(request, listener);
|
||||||
|
}
|
||||||
|
|
||||||
|
@Override public ActionFuture<OpenIndexResponse> open(OpenIndexRequest request) {
|
||||||
|
return openIndexAction.execute(request);
|
||||||
|
}
|
||||||
|
|
||||||
|
@Override public void open(OpenIndexRequest request, ActionListener<OpenIndexResponse> listener) {
|
||||||
|
openIndexAction.execute(request, listener);
|
||||||
|
}
|
||||||
|
|
||||||
@Override public ActionFuture<RefreshResponse> refresh(RefreshRequest request) {
|
@Override public ActionFuture<RefreshResponse> refresh(RefreshRequest request) {
|
||||||
return refreshAction.execute(request);
|
return refreshAction.execute(request);
|
||||||
}
|
}
|
||||||
|
|
|
@ -21,12 +21,14 @@ package org.elasticsearch.client.support;
|
||||||
|
|
||||||
import org.elasticsearch.client.action.admin.indices.alias.IndicesAliasesRequestBuilder;
|
import org.elasticsearch.client.action.admin.indices.alias.IndicesAliasesRequestBuilder;
|
||||||
import org.elasticsearch.client.action.admin.indices.cache.clear.ClearIndicesCacheRequestBuilder;
|
import org.elasticsearch.client.action.admin.indices.cache.clear.ClearIndicesCacheRequestBuilder;
|
||||||
|
import org.elasticsearch.client.action.admin.indices.close.CloseIndexRequestBuilder;
|
||||||
import org.elasticsearch.client.action.admin.indices.create.CreateIndexRequestBuilder;
|
import org.elasticsearch.client.action.admin.indices.create.CreateIndexRequestBuilder;
|
||||||
import org.elasticsearch.client.action.admin.indices.delete.DeleteIndexRequestBuilder;
|
import org.elasticsearch.client.action.admin.indices.delete.DeleteIndexRequestBuilder;
|
||||||
import org.elasticsearch.client.action.admin.indices.flush.FlushRequestBuilder;
|
import org.elasticsearch.client.action.admin.indices.flush.FlushRequestBuilder;
|
||||||
import org.elasticsearch.client.action.admin.indices.gateway.snapshot.GatewaySnapshotRequestBuilder;
|
import org.elasticsearch.client.action.admin.indices.gateway.snapshot.GatewaySnapshotRequestBuilder;
|
||||||
import org.elasticsearch.client.action.admin.indices.mapping.delete.DeleteMappingRequestBuilder;
|
import org.elasticsearch.client.action.admin.indices.mapping.delete.DeleteMappingRequestBuilder;
|
||||||
import org.elasticsearch.client.action.admin.indices.mapping.put.PutMappingRequestBuilder;
|
import org.elasticsearch.client.action.admin.indices.mapping.put.PutMappingRequestBuilder;
|
||||||
|
import org.elasticsearch.client.action.admin.indices.open.OpenIndexRequestBuilder;
|
||||||
import org.elasticsearch.client.action.admin.indices.optimize.OptimizeRequestBuilder;
|
import org.elasticsearch.client.action.admin.indices.optimize.OptimizeRequestBuilder;
|
||||||
import org.elasticsearch.client.action.admin.indices.refresh.RefreshRequestBuilder;
|
import org.elasticsearch.client.action.admin.indices.refresh.RefreshRequestBuilder;
|
||||||
import org.elasticsearch.client.action.admin.indices.settings.UpdateSettingsRequestBuilder;
|
import org.elasticsearch.client.action.admin.indices.settings.UpdateSettingsRequestBuilder;
|
||||||
|
@ -54,6 +56,14 @@ public abstract class AbstractIndicesAdminClient implements InternalIndicesAdmin
|
||||||
return new DeleteIndexRequestBuilder(this, index);
|
return new DeleteIndexRequestBuilder(this, index);
|
||||||
}
|
}
|
||||||
|
|
||||||
|
@Override public CloseIndexRequestBuilder prepareClose(String index) {
|
||||||
|
return new CloseIndexRequestBuilder(this, index);
|
||||||
|
}
|
||||||
|
|
||||||
|
@Override public OpenIndexRequestBuilder prepareOpen(String index) {
|
||||||
|
return new OpenIndexRequestBuilder(this, index);
|
||||||
|
}
|
||||||
|
|
||||||
@Override public FlushRequestBuilder prepareFlush(String... indices) {
|
@Override public FlushRequestBuilder prepareFlush(String... indices) {
|
||||||
return new FlushRequestBuilder(this).setIndices(indices);
|
return new FlushRequestBuilder(this).setIndices(indices);
|
||||||
}
|
}
|
||||||
|
|
|
@ -30,12 +30,14 @@ import org.elasticsearch.client.transport.action.admin.cluster.ping.single.Clien
|
||||||
import org.elasticsearch.client.transport.action.admin.cluster.state.ClientTransportClusterStateAction;
|
import org.elasticsearch.client.transport.action.admin.cluster.state.ClientTransportClusterStateAction;
|
||||||
import org.elasticsearch.client.transport.action.admin.indices.alias.ClientTransportIndicesAliasesAction;
|
import org.elasticsearch.client.transport.action.admin.indices.alias.ClientTransportIndicesAliasesAction;
|
||||||
import org.elasticsearch.client.transport.action.admin.indices.cache.clear.ClientTransportClearIndicesCacheAction;
|
import org.elasticsearch.client.transport.action.admin.indices.cache.clear.ClientTransportClearIndicesCacheAction;
|
||||||
|
import org.elasticsearch.client.transport.action.admin.indices.close.ClientTransportCloseIndexAction;
|
||||||
import org.elasticsearch.client.transport.action.admin.indices.create.ClientTransportCreateIndexAction;
|
import org.elasticsearch.client.transport.action.admin.indices.create.ClientTransportCreateIndexAction;
|
||||||
import org.elasticsearch.client.transport.action.admin.indices.delete.ClientTransportDeleteIndexAction;
|
import org.elasticsearch.client.transport.action.admin.indices.delete.ClientTransportDeleteIndexAction;
|
||||||
import org.elasticsearch.client.transport.action.admin.indices.flush.ClientTransportFlushAction;
|
import org.elasticsearch.client.transport.action.admin.indices.flush.ClientTransportFlushAction;
|
||||||
import org.elasticsearch.client.transport.action.admin.indices.gateway.snapshot.ClientTransportGatewaySnapshotAction;
|
import org.elasticsearch.client.transport.action.admin.indices.gateway.snapshot.ClientTransportGatewaySnapshotAction;
|
||||||
import org.elasticsearch.client.transport.action.admin.indices.mapping.delete.ClientTransportDeleteMappingAction;
|
import org.elasticsearch.client.transport.action.admin.indices.mapping.delete.ClientTransportDeleteMappingAction;
|
||||||
import org.elasticsearch.client.transport.action.admin.indices.mapping.put.ClientTransportPutMappingAction;
|
import org.elasticsearch.client.transport.action.admin.indices.mapping.put.ClientTransportPutMappingAction;
|
||||||
|
import org.elasticsearch.client.transport.action.admin.indices.open.ClientTransportOpenIndexAction;
|
||||||
import org.elasticsearch.client.transport.action.admin.indices.optimize.ClientTransportOptimizeAction;
|
import org.elasticsearch.client.transport.action.admin.indices.optimize.ClientTransportOptimizeAction;
|
||||||
import org.elasticsearch.client.transport.action.admin.indices.refresh.ClientTransportRefreshAction;
|
import org.elasticsearch.client.transport.action.admin.indices.refresh.ClientTransportRefreshAction;
|
||||||
import org.elasticsearch.client.transport.action.admin.indices.settings.ClientTransportUpdateSettingsAction;
|
import org.elasticsearch.client.transport.action.admin.indices.settings.ClientTransportUpdateSettingsAction;
|
||||||
|
@ -71,6 +73,8 @@ public class ClientTransportActionModule extends AbstractModule {
|
||||||
bind(ClientTransportOptimizeAction.class).asEagerSingleton();
|
bind(ClientTransportOptimizeAction.class).asEagerSingleton();
|
||||||
bind(ClientTransportCreateIndexAction.class).asEagerSingleton();
|
bind(ClientTransportCreateIndexAction.class).asEagerSingleton();
|
||||||
bind(ClientTransportDeleteIndexAction.class).asEagerSingleton();
|
bind(ClientTransportDeleteIndexAction.class).asEagerSingleton();
|
||||||
|
bind(ClientTransportCloseIndexAction.class).asEagerSingleton();
|
||||||
|
bind(ClientTransportOpenIndexAction.class).asEagerSingleton();
|
||||||
bind(ClientTransportPutMappingAction.class).asEagerSingleton();
|
bind(ClientTransportPutMappingAction.class).asEagerSingleton();
|
||||||
bind(ClientTransportDeleteMappingAction.class).asEagerSingleton();
|
bind(ClientTransportDeleteMappingAction.class).asEagerSingleton();
|
||||||
bind(ClientTransportGatewaySnapshotAction.class).asEagerSingleton();
|
bind(ClientTransportGatewaySnapshotAction.class).asEagerSingleton();
|
||||||
|
|
|
@ -0,0 +1,42 @@
|
||||||
|
/*
|
||||||
|
* 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.client.transport.action.admin.indices.close;
|
||||||
|
|
||||||
|
import org.elasticsearch.action.TransportActions;
|
||||||
|
import org.elasticsearch.action.admin.indices.close.CloseIndexRequest;
|
||||||
|
import org.elasticsearch.action.admin.indices.close.CloseIndexResponse;
|
||||||
|
import org.elasticsearch.client.transport.action.support.BaseClientTransportAction;
|
||||||
|
import org.elasticsearch.common.inject.Inject;
|
||||||
|
import org.elasticsearch.common.settings.Settings;
|
||||||
|
import org.elasticsearch.transport.TransportService;
|
||||||
|
|
||||||
|
/**
|
||||||
|
* @author kimchy (Shay Banon)
|
||||||
|
*/
|
||||||
|
public class ClientTransportCloseIndexAction extends BaseClientTransportAction<CloseIndexRequest, CloseIndexResponse> {
|
||||||
|
|
||||||
|
@Inject public ClientTransportCloseIndexAction(Settings settings, TransportService transportService) {
|
||||||
|
super(settings, transportService, CloseIndexResponse.class);
|
||||||
|
}
|
||||||
|
|
||||||
|
@Override protected String action() {
|
||||||
|
return TransportActions.Admin.Indices.CLOSE;
|
||||||
|
}
|
||||||
|
}
|
|
@ -0,0 +1,42 @@
|
||||||
|
/*
|
||||||
|
* 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.client.transport.action.admin.indices.open;
|
||||||
|
|
||||||
|
import org.elasticsearch.action.TransportActions;
|
||||||
|
import org.elasticsearch.action.admin.indices.open.OpenIndexRequest;
|
||||||
|
import org.elasticsearch.action.admin.indices.open.OpenIndexResponse;
|
||||||
|
import org.elasticsearch.client.transport.action.support.BaseClientTransportAction;
|
||||||
|
import org.elasticsearch.common.inject.Inject;
|
||||||
|
import org.elasticsearch.common.settings.Settings;
|
||||||
|
import org.elasticsearch.transport.TransportService;
|
||||||
|
|
||||||
|
/**
|
||||||
|
* @author kimchy (Shay Banon)
|
||||||
|
*/
|
||||||
|
public class ClientTransportOpenIndexAction extends BaseClientTransportAction<OpenIndexRequest, OpenIndexResponse> {
|
||||||
|
|
||||||
|
@Inject public ClientTransportOpenIndexAction(Settings settings, TransportService transportService) {
|
||||||
|
super(settings, transportService, OpenIndexResponse.class);
|
||||||
|
}
|
||||||
|
|
||||||
|
@Override protected String action() {
|
||||||
|
return TransportActions.Admin.Indices.OPEN;
|
||||||
|
}
|
||||||
|
}
|
|
@ -26,6 +26,8 @@ import org.elasticsearch.action.admin.indices.alias.IndicesAliasesRequest;
|
||||||
import org.elasticsearch.action.admin.indices.alias.IndicesAliasesResponse;
|
import org.elasticsearch.action.admin.indices.alias.IndicesAliasesResponse;
|
||||||
import org.elasticsearch.action.admin.indices.cache.clear.ClearIndicesCacheRequest;
|
import org.elasticsearch.action.admin.indices.cache.clear.ClearIndicesCacheRequest;
|
||||||
import org.elasticsearch.action.admin.indices.cache.clear.ClearIndicesCacheResponse;
|
import org.elasticsearch.action.admin.indices.cache.clear.ClearIndicesCacheResponse;
|
||||||
|
import org.elasticsearch.action.admin.indices.close.CloseIndexRequest;
|
||||||
|
import org.elasticsearch.action.admin.indices.close.CloseIndexResponse;
|
||||||
import org.elasticsearch.action.admin.indices.create.CreateIndexRequest;
|
import org.elasticsearch.action.admin.indices.create.CreateIndexRequest;
|
||||||
import org.elasticsearch.action.admin.indices.create.CreateIndexResponse;
|
import org.elasticsearch.action.admin.indices.create.CreateIndexResponse;
|
||||||
import org.elasticsearch.action.admin.indices.delete.DeleteIndexRequest;
|
import org.elasticsearch.action.admin.indices.delete.DeleteIndexRequest;
|
||||||
|
@ -38,6 +40,8 @@ import org.elasticsearch.action.admin.indices.mapping.delete.DeleteMappingReques
|
||||||
import org.elasticsearch.action.admin.indices.mapping.delete.DeleteMappingResponse;
|
import org.elasticsearch.action.admin.indices.mapping.delete.DeleteMappingResponse;
|
||||||
import org.elasticsearch.action.admin.indices.mapping.put.PutMappingRequest;
|
import org.elasticsearch.action.admin.indices.mapping.put.PutMappingRequest;
|
||||||
import org.elasticsearch.action.admin.indices.mapping.put.PutMappingResponse;
|
import org.elasticsearch.action.admin.indices.mapping.put.PutMappingResponse;
|
||||||
|
import org.elasticsearch.action.admin.indices.open.OpenIndexRequest;
|
||||||
|
import org.elasticsearch.action.admin.indices.open.OpenIndexResponse;
|
||||||
import org.elasticsearch.action.admin.indices.optimize.OptimizeRequest;
|
import org.elasticsearch.action.admin.indices.optimize.OptimizeRequest;
|
||||||
import org.elasticsearch.action.admin.indices.optimize.OptimizeResponse;
|
import org.elasticsearch.action.admin.indices.optimize.OptimizeResponse;
|
||||||
import org.elasticsearch.action.admin.indices.refresh.RefreshRequest;
|
import org.elasticsearch.action.admin.indices.refresh.RefreshRequest;
|
||||||
|
@ -51,12 +55,14 @@ import org.elasticsearch.client.support.AbstractIndicesAdminClient;
|
||||||
import org.elasticsearch.client.transport.TransportClientNodesService;
|
import org.elasticsearch.client.transport.TransportClientNodesService;
|
||||||
import org.elasticsearch.client.transport.action.admin.indices.alias.ClientTransportIndicesAliasesAction;
|
import org.elasticsearch.client.transport.action.admin.indices.alias.ClientTransportIndicesAliasesAction;
|
||||||
import org.elasticsearch.client.transport.action.admin.indices.cache.clear.ClientTransportClearIndicesCacheAction;
|
import org.elasticsearch.client.transport.action.admin.indices.cache.clear.ClientTransportClearIndicesCacheAction;
|
||||||
|
import org.elasticsearch.client.transport.action.admin.indices.close.ClientTransportCloseIndexAction;
|
||||||
import org.elasticsearch.client.transport.action.admin.indices.create.ClientTransportCreateIndexAction;
|
import org.elasticsearch.client.transport.action.admin.indices.create.ClientTransportCreateIndexAction;
|
||||||
import org.elasticsearch.client.transport.action.admin.indices.delete.ClientTransportDeleteIndexAction;
|
import org.elasticsearch.client.transport.action.admin.indices.delete.ClientTransportDeleteIndexAction;
|
||||||
import org.elasticsearch.client.transport.action.admin.indices.flush.ClientTransportFlushAction;
|
import org.elasticsearch.client.transport.action.admin.indices.flush.ClientTransportFlushAction;
|
||||||
import org.elasticsearch.client.transport.action.admin.indices.gateway.snapshot.ClientTransportGatewaySnapshotAction;
|
import org.elasticsearch.client.transport.action.admin.indices.gateway.snapshot.ClientTransportGatewaySnapshotAction;
|
||||||
import org.elasticsearch.client.transport.action.admin.indices.mapping.delete.ClientTransportDeleteMappingAction;
|
import org.elasticsearch.client.transport.action.admin.indices.mapping.delete.ClientTransportDeleteMappingAction;
|
||||||
import org.elasticsearch.client.transport.action.admin.indices.mapping.put.ClientTransportPutMappingAction;
|
import org.elasticsearch.client.transport.action.admin.indices.mapping.put.ClientTransportPutMappingAction;
|
||||||
|
import org.elasticsearch.client.transport.action.admin.indices.open.ClientTransportOpenIndexAction;
|
||||||
import org.elasticsearch.client.transport.action.admin.indices.optimize.ClientTransportOptimizeAction;
|
import org.elasticsearch.client.transport.action.admin.indices.optimize.ClientTransportOptimizeAction;
|
||||||
import org.elasticsearch.client.transport.action.admin.indices.refresh.ClientTransportRefreshAction;
|
import org.elasticsearch.client.transport.action.admin.indices.refresh.ClientTransportRefreshAction;
|
||||||
import org.elasticsearch.client.transport.action.admin.indices.settings.ClientTransportUpdateSettingsAction;
|
import org.elasticsearch.client.transport.action.admin.indices.settings.ClientTransportUpdateSettingsAction;
|
||||||
|
@ -81,6 +87,10 @@ public class InternalTransportIndicesAdminClient extends AbstractIndicesAdminCli
|
||||||
|
|
||||||
private final ClientTransportDeleteIndexAction deleteIndexAction;
|
private final ClientTransportDeleteIndexAction deleteIndexAction;
|
||||||
|
|
||||||
|
private final ClientTransportCloseIndexAction closeIndexAction;
|
||||||
|
|
||||||
|
private final ClientTransportOpenIndexAction openIndexAction;
|
||||||
|
|
||||||
private final ClientTransportRefreshAction refreshAction;
|
private final ClientTransportRefreshAction refreshAction;
|
||||||
|
|
||||||
private final ClientTransportFlushAction flushAction;
|
private final ClientTransportFlushAction flushAction;
|
||||||
|
@ -102,6 +112,7 @@ public class InternalTransportIndicesAdminClient extends AbstractIndicesAdminCli
|
||||||
@Inject public InternalTransportIndicesAdminClient(Settings settings, TransportClientNodesService nodesService, ThreadPool threadPool,
|
@Inject public InternalTransportIndicesAdminClient(Settings settings, TransportClientNodesService nodesService, ThreadPool threadPool,
|
||||||
ClientTransportIndicesStatusAction indicesStatusAction,
|
ClientTransportIndicesStatusAction indicesStatusAction,
|
||||||
ClientTransportCreateIndexAction createIndexAction, ClientTransportDeleteIndexAction deleteIndexAction,
|
ClientTransportCreateIndexAction createIndexAction, ClientTransportDeleteIndexAction deleteIndexAction,
|
||||||
|
ClientTransportCloseIndexAction closeIndexAction, ClientTransportOpenIndexAction openIndexAction,
|
||||||
ClientTransportRefreshAction refreshAction, ClientTransportFlushAction flushAction, ClientTransportOptimizeAction optimizeAction,
|
ClientTransportRefreshAction refreshAction, ClientTransportFlushAction flushAction, ClientTransportOptimizeAction optimizeAction,
|
||||||
ClientTransportPutMappingAction putMappingAction, ClientTransportDeleteMappingAction deleteMappingAction, ClientTransportGatewaySnapshotAction gatewaySnapshotAction,
|
ClientTransportPutMappingAction putMappingAction, ClientTransportDeleteMappingAction deleteMappingAction, ClientTransportGatewaySnapshotAction gatewaySnapshotAction,
|
||||||
ClientTransportIndicesAliasesAction indicesAliasesAction, ClientTransportClearIndicesCacheAction clearIndicesCacheAction,
|
ClientTransportIndicesAliasesAction indicesAliasesAction, ClientTransportClearIndicesCacheAction clearIndicesCacheAction,
|
||||||
|
@ -111,6 +122,8 @@ public class InternalTransportIndicesAdminClient extends AbstractIndicesAdminCli
|
||||||
this.indicesStatusAction = indicesStatusAction;
|
this.indicesStatusAction = indicesStatusAction;
|
||||||
this.createIndexAction = createIndexAction;
|
this.createIndexAction = createIndexAction;
|
||||||
this.deleteIndexAction = deleteIndexAction;
|
this.deleteIndexAction = deleteIndexAction;
|
||||||
|
this.closeIndexAction = closeIndexAction;
|
||||||
|
this.openIndexAction = openIndexAction;
|
||||||
this.refreshAction = refreshAction;
|
this.refreshAction = refreshAction;
|
||||||
this.flushAction = flushAction;
|
this.flushAction = flushAction;
|
||||||
this.optimizeAction = optimizeAction;
|
this.optimizeAction = optimizeAction;
|
||||||
|
@ -177,6 +190,40 @@ public class InternalTransportIndicesAdminClient extends AbstractIndicesAdminCli
|
||||||
});
|
});
|
||||||
}
|
}
|
||||||
|
|
||||||
|
@Override public ActionFuture<CloseIndexResponse> close(final CloseIndexRequest request) {
|
||||||
|
return nodesService.execute(new TransportClientNodesService.NodeCallback<org.elasticsearch.action.ActionFuture<CloseIndexResponse>>() {
|
||||||
|
@Override public ActionFuture<CloseIndexResponse> doWithNode(DiscoveryNode node) throws ElasticSearchException {
|
||||||
|
return closeIndexAction.execute(node, request);
|
||||||
|
}
|
||||||
|
});
|
||||||
|
}
|
||||||
|
|
||||||
|
@Override public void close(final CloseIndexRequest request, final ActionListener<CloseIndexResponse> listener) {
|
||||||
|
nodesService.execute(new TransportClientNodesService.NodeCallback<Object>() {
|
||||||
|
@Override public Object doWithNode(DiscoveryNode node) throws ElasticSearchException {
|
||||||
|
closeIndexAction.execute(node, request, listener);
|
||||||
|
return null;
|
||||||
|
}
|
||||||
|
});
|
||||||
|
}
|
||||||
|
|
||||||
|
@Override public ActionFuture<OpenIndexResponse> open(final OpenIndexRequest request) {
|
||||||
|
return nodesService.execute(new TransportClientNodesService.NodeCallback<org.elasticsearch.action.ActionFuture<OpenIndexResponse>>() {
|
||||||
|
@Override public ActionFuture<OpenIndexResponse> doWithNode(DiscoveryNode node) throws ElasticSearchException {
|
||||||
|
return openIndexAction.execute(node, request);
|
||||||
|
}
|
||||||
|
});
|
||||||
|
}
|
||||||
|
|
||||||
|
@Override public void open(final OpenIndexRequest request, final ActionListener<OpenIndexResponse> listener) {
|
||||||
|
nodesService.execute(new TransportClientNodesService.NodeCallback<Object>() {
|
||||||
|
@Override public Object doWithNode(DiscoveryNode node) throws ElasticSearchException {
|
||||||
|
openIndexAction.execute(node, request, listener);
|
||||||
|
return null;
|
||||||
|
}
|
||||||
|
});
|
||||||
|
}
|
||||||
|
|
||||||
@Override public ActionFuture<RefreshResponse> refresh(final RefreshRequest request) {
|
@Override public ActionFuture<RefreshResponse> refresh(final RefreshRequest request) {
|
||||||
return nodesService.execute(new TransportClientNodesService.NodeCallback<ActionFuture<RefreshResponse>>() {
|
return nodesService.execute(new TransportClientNodesService.NodeCallback<ActionFuture<RefreshResponse>>() {
|
||||||
@Override public ActionFuture<RefreshResponse> doWithNode(DiscoveryNode node) throws ElasticSearchException {
|
@Override public ActionFuture<RefreshResponse> doWithNode(DiscoveryNode node) throws ElasticSearchException {
|
||||||
|
|
|
@ -55,6 +55,7 @@ public class ClusterModule extends AbstractModule implements SpawnModules {
|
||||||
bind(ClusterService.class).to(InternalClusterService.class).asEagerSingleton();
|
bind(ClusterService.class).to(InternalClusterService.class).asEagerSingleton();
|
||||||
bind(MetaDataCreateIndexService.class).asEagerSingleton();
|
bind(MetaDataCreateIndexService.class).asEagerSingleton();
|
||||||
bind(MetaDataDeleteIndexService.class).asEagerSingleton();
|
bind(MetaDataDeleteIndexService.class).asEagerSingleton();
|
||||||
|
bind(MetaDataStateIndexService.class).asEagerSingleton();
|
||||||
bind(MetaDataMappingService.class).asEagerSingleton();
|
bind(MetaDataMappingService.class).asEagerSingleton();
|
||||||
bind(MetaDataIndexAliasesService.class).asEagerSingleton();
|
bind(MetaDataIndexAliasesService.class).asEagerSingleton();
|
||||||
bind(MetaDataUpdateSettingsService.class).asEagerSingleton();
|
bind(MetaDataUpdateSettingsService.class).asEagerSingleton();
|
||||||
|
|
|
@ -19,6 +19,7 @@
|
||||||
|
|
||||||
package org.elasticsearch.cluster.metadata;
|
package org.elasticsearch.cluster.metadata;
|
||||||
|
|
||||||
|
import org.elasticsearch.ElasticSearchIllegalStateException;
|
||||||
import org.elasticsearch.common.Preconditions;
|
import org.elasticsearch.common.Preconditions;
|
||||||
import org.elasticsearch.common.collect.ImmutableMap;
|
import org.elasticsearch.common.collect.ImmutableMap;
|
||||||
import org.elasticsearch.common.collect.ImmutableSet;
|
import org.elasticsearch.common.collect.ImmutableSet;
|
||||||
|
@ -46,12 +47,47 @@ import static org.elasticsearch.common.settings.ImmutableSettings.*;
|
||||||
@Immutable
|
@Immutable
|
||||||
public class IndexMetaData {
|
public class IndexMetaData {
|
||||||
|
|
||||||
|
public static enum State {
|
||||||
|
OPEN((byte) 0),
|
||||||
|
CLOSE((byte) 1);
|
||||||
|
|
||||||
|
private final byte id;
|
||||||
|
|
||||||
|
State(byte id) {
|
||||||
|
this.id = id;
|
||||||
|
}
|
||||||
|
|
||||||
|
public byte id() {
|
||||||
|
return this.id;
|
||||||
|
}
|
||||||
|
|
||||||
|
public static State fromId(byte id) {
|
||||||
|
if (id == 0) {
|
||||||
|
return OPEN;
|
||||||
|
} else if (id == 1) {
|
||||||
|
return CLOSE;
|
||||||
|
}
|
||||||
|
throw new ElasticSearchIllegalStateException("No state match for id [" + id + "]");
|
||||||
|
}
|
||||||
|
|
||||||
|
public static State fromString(String state) {
|
||||||
|
if ("open".equals(state)) {
|
||||||
|
return OPEN;
|
||||||
|
} else if ("close".equals(state)) {
|
||||||
|
return CLOSE;
|
||||||
|
}
|
||||||
|
throw new ElasticSearchIllegalStateException("No state match for [" + state + "]");
|
||||||
|
}
|
||||||
|
}
|
||||||
|
|
||||||
public static final String SETTING_NUMBER_OF_SHARDS = "index.number_of_shards";
|
public static final String SETTING_NUMBER_OF_SHARDS = "index.number_of_shards";
|
||||||
|
|
||||||
public static final String SETTING_NUMBER_OF_REPLICAS = "index.number_of_replicas";
|
public static final String SETTING_NUMBER_OF_REPLICAS = "index.number_of_replicas";
|
||||||
|
|
||||||
private final String index;
|
private final String index;
|
||||||
|
|
||||||
|
private final State state;
|
||||||
|
|
||||||
private final ImmutableSet<String> aliases;
|
private final ImmutableSet<String> aliases;
|
||||||
|
|
||||||
private final Settings settings;
|
private final Settings settings;
|
||||||
|
@ -60,10 +96,11 @@ public class IndexMetaData {
|
||||||
|
|
||||||
private transient final int totalNumberOfShards;
|
private transient final int totalNumberOfShards;
|
||||||
|
|
||||||
private IndexMetaData(String index, Settings settings, ImmutableMap<String, CompressedString> mappings) {
|
private IndexMetaData(String index, State state, Settings settings, ImmutableMap<String, CompressedString> mappings) {
|
||||||
Preconditions.checkArgument(settings.getAsInt(SETTING_NUMBER_OF_SHARDS, -1) != -1, "must specify numberOfShards for index [" + index + "]");
|
Preconditions.checkArgument(settings.getAsInt(SETTING_NUMBER_OF_SHARDS, -1) != -1, "must specify numberOfShards for index [" + index + "]");
|
||||||
Preconditions.checkArgument(settings.getAsInt(SETTING_NUMBER_OF_REPLICAS, -1) != -1, "must specify numberOfReplicas for index [" + index + "]");
|
Preconditions.checkArgument(settings.getAsInt(SETTING_NUMBER_OF_REPLICAS, -1) != -1, "must specify numberOfReplicas for index [" + index + "]");
|
||||||
this.index = index;
|
this.index = index;
|
||||||
|
this.state = state;
|
||||||
this.settings = settings;
|
this.settings = settings;
|
||||||
this.mappings = mappings;
|
this.mappings = mappings;
|
||||||
this.totalNumberOfShards = numberOfShards() * (numberOfReplicas() + 1);
|
this.totalNumberOfShards = numberOfShards() * (numberOfReplicas() + 1);
|
||||||
|
@ -79,6 +116,14 @@ public class IndexMetaData {
|
||||||
return index();
|
return index();
|
||||||
}
|
}
|
||||||
|
|
||||||
|
public State state() {
|
||||||
|
return this.state;
|
||||||
|
}
|
||||||
|
|
||||||
|
public State getState() {
|
||||||
|
return state();
|
||||||
|
}
|
||||||
|
|
||||||
public int numberOfShards() {
|
public int numberOfShards() {
|
||||||
return settings.getAsInt(SETTING_NUMBER_OF_SHARDS, -1);
|
return settings.getAsInt(SETTING_NUMBER_OF_SHARDS, -1);
|
||||||
}
|
}
|
||||||
|
@ -143,6 +188,8 @@ public class IndexMetaData {
|
||||||
|
|
||||||
private String index;
|
private String index;
|
||||||
|
|
||||||
|
private State state = State.OPEN;
|
||||||
|
|
||||||
private Settings settings = ImmutableSettings.Builder.EMPTY_SETTINGS;
|
private Settings settings = ImmutableSettings.Builder.EMPTY_SETTINGS;
|
||||||
|
|
||||||
private MapBuilder<String, CompressedString> mappings = MapBuilder.newMapBuilder();
|
private MapBuilder<String, CompressedString> mappings = MapBuilder.newMapBuilder();
|
||||||
|
@ -155,6 +202,7 @@ public class IndexMetaData {
|
||||||
this(indexMetaData.index());
|
this(indexMetaData.index());
|
||||||
settings(indexMetaData.settings());
|
settings(indexMetaData.settings());
|
||||||
mappings.putAll(indexMetaData.mappings);
|
mappings.putAll(indexMetaData.mappings);
|
||||||
|
this.state = indexMetaData.state;
|
||||||
}
|
}
|
||||||
|
|
||||||
public String index() {
|
public String index() {
|
||||||
|
@ -204,13 +252,20 @@ public class IndexMetaData {
|
||||||
return this;
|
return this;
|
||||||
}
|
}
|
||||||
|
|
||||||
|
public Builder state(State state) {
|
||||||
|
this.state = state;
|
||||||
|
return this;
|
||||||
|
}
|
||||||
|
|
||||||
public IndexMetaData build() {
|
public IndexMetaData build() {
|
||||||
return new IndexMetaData(index, settings, mappings.immutableMap());
|
return new IndexMetaData(index, state, settings, mappings.immutableMap());
|
||||||
}
|
}
|
||||||
|
|
||||||
public static void toXContent(IndexMetaData indexMetaData, XContentBuilder builder, ToXContent.Params params) throws IOException {
|
public static void toXContent(IndexMetaData indexMetaData, XContentBuilder builder, ToXContent.Params params) throws IOException {
|
||||||
builder.startObject(indexMetaData.index());
|
builder.startObject(indexMetaData.index());
|
||||||
|
|
||||||
|
builder.field("state", indexMetaData.state().toString().toLowerCase());
|
||||||
|
|
||||||
builder.startObject("settings");
|
builder.startObject("settings");
|
||||||
for (Map.Entry<String, String> entry : indexMetaData.settings().getAsMap().entrySet()) {
|
for (Map.Entry<String, String> entry : indexMetaData.settings().getAsMap().entrySet()) {
|
||||||
builder.field(entry.getKey(), entry.getValue());
|
builder.field(entry.getKey(), entry.getValue());
|
||||||
|
@ -262,6 +317,10 @@ public class IndexMetaData {
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
} else if (token.isValue()) {
|
||||||
|
if ("state".equals(currentFieldName)) {
|
||||||
|
builder.state(State.fromString(parser.text()));
|
||||||
|
}
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
return builder.build();
|
return builder.build();
|
||||||
|
@ -269,6 +328,7 @@ public class IndexMetaData {
|
||||||
|
|
||||||
public static IndexMetaData readFrom(StreamInput in, Settings globalSettings) throws IOException {
|
public static IndexMetaData readFrom(StreamInput in, Settings globalSettings) throws IOException {
|
||||||
Builder builder = new Builder(in.readUTF());
|
Builder builder = new Builder(in.readUTF());
|
||||||
|
builder.state(State.fromId(in.readByte()));
|
||||||
builder.settings(readSettingsFromStream(in, globalSettings));
|
builder.settings(readSettingsFromStream(in, globalSettings));
|
||||||
int mappingsSize = in.readVInt();
|
int mappingsSize = in.readVInt();
|
||||||
for (int i = 0; i < mappingsSize; i++) {
|
for (int i = 0; i < mappingsSize; i++) {
|
||||||
|
@ -279,6 +339,7 @@ public class IndexMetaData {
|
||||||
|
|
||||||
public static void writeTo(IndexMetaData indexMetaData, StreamOutput out) throws IOException {
|
public static void writeTo(IndexMetaData indexMetaData, StreamOutput out) throws IOException {
|
||||||
out.writeUTF(indexMetaData.index());
|
out.writeUTF(indexMetaData.index());
|
||||||
|
out.writeByte(indexMetaData.state().id());
|
||||||
writeSettingsToStream(indexMetaData.settings(), out);
|
writeSettingsToStream(indexMetaData.settings(), out);
|
||||||
out.writeVInt(indexMetaData.mappings().size());
|
out.writeVInt(indexMetaData.mappings().size());
|
||||||
for (Map.Entry<String, CompressedString> entry : indexMetaData.mappings().entrySet()) {
|
for (Map.Entry<String, CompressedString> entry : indexMetaData.mappings().entrySet()) {
|
||||||
|
|
|
@ -215,6 +215,11 @@ public class MetaData implements Iterable<IndexMetaData> {
|
||||||
return indices.values().iterator();
|
return indices.values().iterator();
|
||||||
}
|
}
|
||||||
|
|
||||||
|
public static Builder builder() {
|
||||||
|
return new Builder();
|
||||||
|
}
|
||||||
|
|
||||||
|
|
||||||
public static Builder newMetaDataBuilder() {
|
public static Builder newMetaDataBuilder() {
|
||||||
return new Builder();
|
return new Builder();
|
||||||
}
|
}
|
||||||
|
|
|
@ -195,6 +195,7 @@ public class MetaDataCreateIndexService extends AbstractComponent {
|
||||||
for (Map.Entry<String, CompressedString> entry : mappings.entrySet()) {
|
for (Map.Entry<String, CompressedString> entry : mappings.entrySet()) {
|
||||||
indexMetaDataBuilder.putMapping(entry.getKey(), entry.getValue());
|
indexMetaDataBuilder.putMapping(entry.getKey(), entry.getValue());
|
||||||
}
|
}
|
||||||
|
indexMetaDataBuilder.state(request.state);
|
||||||
final IndexMetaData indexMetaData = indexMetaDataBuilder.build();
|
final IndexMetaData indexMetaData = indexMetaDataBuilder.build();
|
||||||
|
|
||||||
MetaData newMetaData = newMetaDataBuilder()
|
MetaData newMetaData = newMetaDataBuilder()
|
||||||
|
@ -210,6 +211,9 @@ public class MetaDataCreateIndexService extends AbstractComponent {
|
||||||
blocks.addIndexBlock(request.index, block);
|
blocks.addIndexBlock(request.index, block);
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
if (request.state == State.CLOSE) {
|
||||||
|
blocks.addIndexBlock(request.index, MetaDataStateIndexService.INDEX_CLOSED_BLOCK);
|
||||||
|
}
|
||||||
|
|
||||||
return newClusterStateBuilder().state(currentState).blocks(blocks).metaData(newMetaData).build();
|
return newClusterStateBuilder().state(currentState).blocks(blocks).metaData(newMetaData).build();
|
||||||
} catch (Exception e) {
|
} catch (Exception e) {
|
||||||
|
@ -219,12 +223,12 @@ public class MetaDataCreateIndexService extends AbstractComponent {
|
||||||
}
|
}
|
||||||
|
|
||||||
@Override public void clusterStateProcessed(ClusterState clusterState) {
|
@Override public void clusterStateProcessed(ClusterState clusterState) {
|
||||||
|
if (request.state == State.CLOSE) { // no need to do shard allocated when closed...
|
||||||
|
return;
|
||||||
|
}
|
||||||
clusterService.submitStateUpdateTask("reroute after index [" + request.index + "] creation", new ProcessedClusterStateUpdateTask() {
|
clusterService.submitStateUpdateTask("reroute after index [" + request.index + "] creation", new ProcessedClusterStateUpdateTask() {
|
||||||
@Override public ClusterState execute(ClusterState currentState) {
|
@Override public ClusterState execute(ClusterState currentState) {
|
||||||
RoutingTable.Builder routingTableBuilder = new RoutingTable.Builder();
|
RoutingTable.Builder routingTableBuilder = RoutingTable.builder().routingTable(currentState.routingTable());
|
||||||
for (IndexRoutingTable indexRoutingTable : currentState.routingTable().indicesRouting().values()) {
|
|
||||||
routingTableBuilder.add(indexRoutingTable);
|
|
||||||
}
|
|
||||||
IndexRoutingTable.Builder indexRoutingBuilder = new IndexRoutingTable.Builder(request.index)
|
IndexRoutingTable.Builder indexRoutingBuilder = new IndexRoutingTable.Builder(request.index)
|
||||||
.initializeEmpty(currentState.metaData().index(request.index));
|
.initializeEmpty(currentState.metaData().index(request.index));
|
||||||
routingTableBuilder.add(indexRoutingBuilder);
|
routingTableBuilder.add(indexRoutingBuilder);
|
||||||
|
@ -270,6 +274,8 @@ public class MetaDataCreateIndexService extends AbstractComponent {
|
||||||
|
|
||||||
final String index;
|
final String index;
|
||||||
|
|
||||||
|
State state = State.OPEN;
|
||||||
|
|
||||||
Settings settings = ImmutableSettings.Builder.EMPTY_SETTINGS;
|
Settings settings = ImmutableSettings.Builder.EMPTY_SETTINGS;
|
||||||
|
|
||||||
Map<String, String> mappings = Maps.newHashMap();
|
Map<String, String> mappings = Maps.newHashMap();
|
||||||
|
@ -305,6 +311,11 @@ public class MetaDataCreateIndexService extends AbstractComponent {
|
||||||
return this;
|
return this;
|
||||||
}
|
}
|
||||||
|
|
||||||
|
public Request state(State state) {
|
||||||
|
this.state = state;
|
||||||
|
return this;
|
||||||
|
}
|
||||||
|
|
||||||
public Request timeout(TimeValue timeout) {
|
public Request timeout(TimeValue timeout) {
|
||||||
this.timeout = timeout;
|
this.timeout = timeout;
|
||||||
return this;
|
return this;
|
||||||
|
|
|
@ -0,0 +1,162 @@
|
||||||
|
/*
|
||||||
|
* 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.cluster.metadata;
|
||||||
|
|
||||||
|
import org.elasticsearch.cluster.ClusterService;
|
||||||
|
import org.elasticsearch.cluster.ClusterState;
|
||||||
|
import org.elasticsearch.cluster.ProcessedClusterStateUpdateTask;
|
||||||
|
import org.elasticsearch.cluster.block.ClusterBlock;
|
||||||
|
import org.elasticsearch.cluster.block.ClusterBlockLevel;
|
||||||
|
import org.elasticsearch.cluster.block.ClusterBlocks;
|
||||||
|
import org.elasticsearch.cluster.routing.IndexRoutingTable;
|
||||||
|
import org.elasticsearch.cluster.routing.RoutingTable;
|
||||||
|
import org.elasticsearch.common.component.AbstractComponent;
|
||||||
|
import org.elasticsearch.common.inject.Inject;
|
||||||
|
import org.elasticsearch.common.settings.Settings;
|
||||||
|
import org.elasticsearch.common.unit.TimeValue;
|
||||||
|
import org.elasticsearch.gateway.GatewayService;
|
||||||
|
import org.elasticsearch.index.Index;
|
||||||
|
import org.elasticsearch.indices.IndexMissingException;
|
||||||
|
|
||||||
|
/**
|
||||||
|
* @author kimchy (shay.banon)
|
||||||
|
*/
|
||||||
|
public class MetaDataStateIndexService extends AbstractComponent {
|
||||||
|
|
||||||
|
public static final ClusterBlock INDEX_CLOSED_BLOCK = new ClusterBlock(4, "index closed", ClusterBlockLevel.READ_WRITE);
|
||||||
|
|
||||||
|
private final ClusterService clusterService;
|
||||||
|
|
||||||
|
@Inject public MetaDataStateIndexService(Settings settings, ClusterService clusterService) {
|
||||||
|
super(settings);
|
||||||
|
this.clusterService = clusterService;
|
||||||
|
}
|
||||||
|
|
||||||
|
public void closeIndex(final Request request, final Listener listener) {
|
||||||
|
clusterService.submitStateUpdateTask("close-index [" + request.index + "]", new ProcessedClusterStateUpdateTask() {
|
||||||
|
@Override public ClusterState execute(ClusterState currentState) {
|
||||||
|
|
||||||
|
IndexMetaData indexMetaData = currentState.metaData().index(request.index);
|
||||||
|
if (indexMetaData == null) {
|
||||||
|
listener.onFailure(new IndexMissingException(new Index(request.index)));
|
||||||
|
return currentState;
|
||||||
|
}
|
||||||
|
|
||||||
|
if (indexMetaData.state() == IndexMetaData.State.CLOSE) {
|
||||||
|
listener.onResponse(new Response(true));
|
||||||
|
return currentState;
|
||||||
|
}
|
||||||
|
|
||||||
|
logger.info("[{}] closing index", request.index);
|
||||||
|
|
||||||
|
MetaData.Builder mdBuilder = MetaData.builder()
|
||||||
|
.metaData(currentState.metaData())
|
||||||
|
.put(IndexMetaData.newIndexMetaDataBuilder(currentState.metaData().index(request.index)).state(IndexMetaData.State.CLOSE));
|
||||||
|
|
||||||
|
RoutingTable.Builder rtBuilder = RoutingTable.builder()
|
||||||
|
.routingTable(currentState.routingTable())
|
||||||
|
.remove(request.index);
|
||||||
|
|
||||||
|
ClusterBlocks.Builder blocks = ClusterBlocks.builder().blocks(currentState.blocks())
|
||||||
|
.addIndexBlock(request.index, INDEX_CLOSED_BLOCK);
|
||||||
|
|
||||||
|
return ClusterState.builder().state(currentState).metaData(mdBuilder).routingTable(rtBuilder).blocks(blocks).build();
|
||||||
|
}
|
||||||
|
|
||||||
|
@Override public void clusterStateProcessed(ClusterState clusterState) {
|
||||||
|
listener.onResponse(new Response(true));
|
||||||
|
}
|
||||||
|
});
|
||||||
|
}
|
||||||
|
|
||||||
|
public void openIndex(final Request request, final Listener listener) {
|
||||||
|
clusterService.submitStateUpdateTask("open-index [" + request.index + "]", new ProcessedClusterStateUpdateTask() {
|
||||||
|
@Override public ClusterState execute(ClusterState currentState) {
|
||||||
|
|
||||||
|
IndexMetaData indexMetaData = currentState.metaData().index(request.index);
|
||||||
|
if (indexMetaData == null) {
|
||||||
|
listener.onFailure(new IndexMissingException(new Index(request.index)));
|
||||||
|
return currentState;
|
||||||
|
}
|
||||||
|
|
||||||
|
if (indexMetaData.state() == IndexMetaData.State.OPEN) {
|
||||||
|
listener.onResponse(new Response(true));
|
||||||
|
return currentState;
|
||||||
|
}
|
||||||
|
|
||||||
|
logger.info("[{}] opening index", request.index);
|
||||||
|
|
||||||
|
MetaData.Builder mdBuilder = MetaData.builder()
|
||||||
|
.metaData(currentState.metaData())
|
||||||
|
.put(IndexMetaData.newIndexMetaDataBuilder(currentState.metaData().index(request.index)).state(IndexMetaData.State.OPEN));
|
||||||
|
|
||||||
|
RoutingTable.Builder rtBuilder = RoutingTable.builder().routingTable(currentState.routingTable());
|
||||||
|
IndexRoutingTable.Builder indexRoutingBuilder = new IndexRoutingTable.Builder(request.index)
|
||||||
|
.initializeEmpty(currentState.metaData().index(request.index));
|
||||||
|
rtBuilder.add(indexRoutingBuilder);
|
||||||
|
|
||||||
|
ClusterBlocks.Builder blocks = ClusterBlocks.builder().blocks(currentState.blocks())
|
||||||
|
.removeIndexBlock(request.index, INDEX_CLOSED_BLOCK)
|
||||||
|
.addIndexBlock(request.index, GatewayService.INDEX_NOT_RECOVERED_BLOCK);
|
||||||
|
|
||||||
|
return ClusterState.builder().state(currentState).metaData(mdBuilder).routingTable(rtBuilder).blocks(blocks).build();
|
||||||
|
}
|
||||||
|
|
||||||
|
@Override public void clusterStateProcessed(ClusterState clusterState) {
|
||||||
|
listener.onResponse(new Response(true));
|
||||||
|
}
|
||||||
|
});
|
||||||
|
}
|
||||||
|
|
||||||
|
public static interface Listener {
|
||||||
|
|
||||||
|
void onResponse(Response response);
|
||||||
|
|
||||||
|
void onFailure(Throwable t);
|
||||||
|
}
|
||||||
|
|
||||||
|
public static class Request {
|
||||||
|
|
||||||
|
final String index;
|
||||||
|
|
||||||
|
TimeValue timeout = TimeValue.timeValueSeconds(10);
|
||||||
|
|
||||||
|
public Request(String index) {
|
||||||
|
this.index = index;
|
||||||
|
}
|
||||||
|
|
||||||
|
public Request timeout(TimeValue timeout) {
|
||||||
|
this.timeout = timeout;
|
||||||
|
return this;
|
||||||
|
}
|
||||||
|
}
|
||||||
|
|
||||||
|
public static class Response {
|
||||||
|
private final boolean acknowledged;
|
||||||
|
|
||||||
|
public Response(boolean acknowledged) {
|
||||||
|
this.acknowledged = acknowledged;
|
||||||
|
}
|
||||||
|
|
||||||
|
public boolean acknowledged() {
|
||||||
|
return acknowledged;
|
||||||
|
}
|
||||||
|
}
|
||||||
|
}
|
|
@ -150,7 +150,9 @@ public class RoutingTable implements Iterable<IndexRoutingTable> {
|
||||||
for (String index : indices) {
|
for (String index : indices) {
|
||||||
IndexRoutingTable indexRoutingTable = index(index);
|
IndexRoutingTable indexRoutingTable = index(index);
|
||||||
if (indexRoutingTable == null) {
|
if (indexRoutingTable == null) {
|
||||||
throw new IndexMissingException(new Index(index));
|
continue;
|
||||||
|
// we simply ignore indices that don't exists (make sense for operations that use it currently)
|
||||||
|
// throw new IndexMissingException(new Index(index));
|
||||||
}
|
}
|
||||||
for (IndexShardRoutingTable indexShardRoutingTable : indexRoutingTable) {
|
for (IndexShardRoutingTable indexShardRoutingTable : indexRoutingTable) {
|
||||||
for (ShardRouting shardRouting : indexShardRoutingTable) {
|
for (ShardRouting shardRouting : indexShardRoutingTable) {
|
||||||
|
@ -188,6 +190,10 @@ public class RoutingTable implements Iterable<IndexRoutingTable> {
|
||||||
return its;
|
return its;
|
||||||
}
|
}
|
||||||
|
|
||||||
|
public static Builder builder() {
|
||||||
|
return new Builder();
|
||||||
|
}
|
||||||
|
|
||||||
public static Builder newRoutingTableBuilder() {
|
public static Builder newRoutingTableBuilder() {
|
||||||
return new Builder();
|
return new Builder();
|
||||||
}
|
}
|
||||||
|
@ -249,6 +255,11 @@ public class RoutingTable implements Iterable<IndexRoutingTable> {
|
||||||
return this;
|
return this;
|
||||||
}
|
}
|
||||||
|
|
||||||
|
public Builder remove(String index) {
|
||||||
|
indicesRouting.remove(index);
|
||||||
|
return this;
|
||||||
|
}
|
||||||
|
|
||||||
public Builder updateNodes(RoutingNodes routingNodes) {
|
public Builder updateNodes(RoutingNodes routingNodes) {
|
||||||
Map<String, IndexRoutingTable.Builder> indexRoutingTableBuilders = newHashMap();
|
Map<String, IndexRoutingTable.Builder> indexRoutingTableBuilders = newHashMap();
|
||||||
for (RoutingNode routingNode : routingNodes) {
|
for (RoutingNode routingNode : routingNodes) {
|
||||||
|
|
|
@ -180,6 +180,7 @@ public class LocalGateway extends AbstractLifecycleComponent<Gateway> implements
|
||||||
createIndexService.createIndex(new MetaDataCreateIndexService.Request("gateway", indexMetaData.index())
|
createIndexService.createIndex(new MetaDataCreateIndexService.Request("gateway", indexMetaData.index())
|
||||||
.settings(indexMetaData.settings())
|
.settings(indexMetaData.settings())
|
||||||
.mappingsCompressed(indexMetaData.mappings())
|
.mappingsCompressed(indexMetaData.mappings())
|
||||||
|
.state(indexMetaData.state())
|
||||||
.blocks(ImmutableSet.of(GatewayService.INDEX_NOT_RECOVERED_BLOCK))
|
.blocks(ImmutableSet.of(GatewayService.INDEX_NOT_RECOVERED_BLOCK))
|
||||||
.timeout(timeValueSeconds(30)),
|
.timeout(timeValueSeconds(30)),
|
||||||
|
|
||||||
|
|
|
@ -150,6 +150,7 @@ public abstract class SharedStorageGateway extends AbstractLifecycleComponent<Ga
|
||||||
createIndexService.createIndex(new MetaDataCreateIndexService.Request("gateway", indexMetaData.index())
|
createIndexService.createIndex(new MetaDataCreateIndexService.Request("gateway", indexMetaData.index())
|
||||||
.settings(indexMetaData.settings())
|
.settings(indexMetaData.settings())
|
||||||
.mappingsCompressed(indexMetaData.mappings())
|
.mappingsCompressed(indexMetaData.mappings())
|
||||||
|
.state(indexMetaData.state())
|
||||||
.blocks(ImmutableSet.of(GatewayService.INDEX_NOT_RECOVERED_BLOCK))
|
.blocks(ImmutableSet.of(GatewayService.INDEX_NOT_RECOVERED_BLOCK))
|
||||||
.timeout(timeValueSeconds(30)),
|
.timeout(timeValueSeconds(30)),
|
||||||
|
|
||||||
|
|
|
@ -31,6 +31,7 @@ import org.elasticsearch.rest.action.admin.cluster.ping.single.RestSinglePingAct
|
||||||
import org.elasticsearch.rest.action.admin.cluster.state.RestClusterStateAction;
|
import org.elasticsearch.rest.action.admin.cluster.state.RestClusterStateAction;
|
||||||
import org.elasticsearch.rest.action.admin.indices.alias.RestIndicesAliasesAction;
|
import org.elasticsearch.rest.action.admin.indices.alias.RestIndicesAliasesAction;
|
||||||
import org.elasticsearch.rest.action.admin.indices.cache.clear.RestClearIndicesCacheAction;
|
import org.elasticsearch.rest.action.admin.indices.cache.clear.RestClearIndicesCacheAction;
|
||||||
|
import org.elasticsearch.rest.action.admin.indices.close.RestCloseIndexAction;
|
||||||
import org.elasticsearch.rest.action.admin.indices.create.RestCreateIndexAction;
|
import org.elasticsearch.rest.action.admin.indices.create.RestCreateIndexAction;
|
||||||
import org.elasticsearch.rest.action.admin.indices.delete.RestDeleteIndexAction;
|
import org.elasticsearch.rest.action.admin.indices.delete.RestDeleteIndexAction;
|
||||||
import org.elasticsearch.rest.action.admin.indices.flush.RestFlushAction;
|
import org.elasticsearch.rest.action.admin.indices.flush.RestFlushAction;
|
||||||
|
@ -38,6 +39,7 @@ import org.elasticsearch.rest.action.admin.indices.gateway.snapshot.RestGatewayS
|
||||||
import org.elasticsearch.rest.action.admin.indices.mapping.delete.RestDeleteMappingAction;
|
import org.elasticsearch.rest.action.admin.indices.mapping.delete.RestDeleteMappingAction;
|
||||||
import org.elasticsearch.rest.action.admin.indices.mapping.get.RestGetMappingAction;
|
import org.elasticsearch.rest.action.admin.indices.mapping.get.RestGetMappingAction;
|
||||||
import org.elasticsearch.rest.action.admin.indices.mapping.put.RestPutMappingAction;
|
import org.elasticsearch.rest.action.admin.indices.mapping.put.RestPutMappingAction;
|
||||||
|
import org.elasticsearch.rest.action.admin.indices.open.RestOpenIndexAction;
|
||||||
import org.elasticsearch.rest.action.admin.indices.optimize.RestOptimizeAction;
|
import org.elasticsearch.rest.action.admin.indices.optimize.RestOptimizeAction;
|
||||||
import org.elasticsearch.rest.action.admin.indices.refresh.RestRefreshAction;
|
import org.elasticsearch.rest.action.admin.indices.refresh.RestRefreshAction;
|
||||||
import org.elasticsearch.rest.action.admin.indices.settings.RestUpdateSettingsAction;
|
import org.elasticsearch.rest.action.admin.indices.settings.RestUpdateSettingsAction;
|
||||||
|
@ -76,6 +78,8 @@ public class RestActionModule extends AbstractModule {
|
||||||
bind(RestIndicesAliasesAction.class).asEagerSingleton();
|
bind(RestIndicesAliasesAction.class).asEagerSingleton();
|
||||||
bind(RestCreateIndexAction.class).asEagerSingleton();
|
bind(RestCreateIndexAction.class).asEagerSingleton();
|
||||||
bind(RestDeleteIndexAction.class).asEagerSingleton();
|
bind(RestDeleteIndexAction.class).asEagerSingleton();
|
||||||
|
bind(RestCloseIndexAction.class).asEagerSingleton();
|
||||||
|
bind(RestOpenIndexAction.class).asEagerSingleton();
|
||||||
bind(RestUpdateSettingsAction.class).asEagerSingleton();
|
bind(RestUpdateSettingsAction.class).asEagerSingleton();
|
||||||
|
|
||||||
bind(RestPutMappingAction.class).asEagerSingleton();
|
bind(RestPutMappingAction.class).asEagerSingleton();
|
||||||
|
|
|
@ -140,6 +140,8 @@ public class RestClusterStateAction extends BaseRestHandler {
|
||||||
for (IndexMetaData indexMetaData : state.metaData()) {
|
for (IndexMetaData indexMetaData : state.metaData()) {
|
||||||
builder.startObject(indexMetaData.index(), XContentBuilder.FieldCaseConversion.NONE);
|
builder.startObject(indexMetaData.index(), XContentBuilder.FieldCaseConversion.NONE);
|
||||||
|
|
||||||
|
builder.field("state", indexMetaData.state().toString().toLowerCase());
|
||||||
|
|
||||||
builder.startObject("settings");
|
builder.startObject("settings");
|
||||||
Settings settings = settingsFilter.filterSettings(indexMetaData.settings());
|
Settings settings = settingsFilter.filterSettings(indexMetaData.settings());
|
||||||
for (Map.Entry<String, String> entry : settings.getAsMap().entrySet()) {
|
for (Map.Entry<String, String> entry : settings.getAsMap().entrySet()) {
|
||||||
|
|
|
@ -0,0 +1,87 @@
|
||||||
|
/*
|
||||||
|
* 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.rest.action.admin.indices.close;
|
||||||
|
|
||||||
|
import org.elasticsearch.action.ActionListener;
|
||||||
|
import org.elasticsearch.action.admin.indices.close.CloseIndexRequest;
|
||||||
|
import org.elasticsearch.action.admin.indices.close.CloseIndexResponse;
|
||||||
|
import org.elasticsearch.client.Client;
|
||||||
|
import org.elasticsearch.common.inject.Inject;
|
||||||
|
import org.elasticsearch.common.settings.Settings;
|
||||||
|
import org.elasticsearch.common.xcontent.XContentBuilder;
|
||||||
|
import org.elasticsearch.common.xcontent.XContentBuilderString;
|
||||||
|
import org.elasticsearch.indices.IndexMissingException;
|
||||||
|
import org.elasticsearch.rest.*;
|
||||||
|
import org.elasticsearch.rest.action.support.RestXContentBuilder;
|
||||||
|
|
||||||
|
import java.io.IOException;
|
||||||
|
|
||||||
|
import static org.elasticsearch.ExceptionsHelper.*;
|
||||||
|
import static org.elasticsearch.common.unit.TimeValue.*;
|
||||||
|
import static org.elasticsearch.rest.RestResponse.Status.*;
|
||||||
|
|
||||||
|
/**
|
||||||
|
* @author kimchy (shay.banon)
|
||||||
|
*/
|
||||||
|
public class RestCloseIndexAction extends BaseRestHandler {
|
||||||
|
|
||||||
|
@Inject public RestCloseIndexAction(Settings settings, Client client, RestController controller) {
|
||||||
|
super(settings, client);
|
||||||
|
controller.registerHandler(RestRequest.Method.POST, "/{index}/_close", this);
|
||||||
|
}
|
||||||
|
|
||||||
|
@Override public void handleRequest(final RestRequest request, final RestChannel channel) {
|
||||||
|
CloseIndexRequest closeIndexRequest = new CloseIndexRequest(request.param("index"));
|
||||||
|
closeIndexRequest.timeout(request.paramAsTime("timeout", timeValueSeconds(10)));
|
||||||
|
client.admin().indices().close(closeIndexRequest, new ActionListener<CloseIndexResponse>() {
|
||||||
|
@Override public void onResponse(CloseIndexResponse response) {
|
||||||
|
try {
|
||||||
|
XContentBuilder builder = RestXContentBuilder.restContentBuilder(request);
|
||||||
|
builder.startObject()
|
||||||
|
.field(Fields.OK, true)
|
||||||
|
.field(Fields.ACKNOWLEDGED, response.acknowledged())
|
||||||
|
.endObject();
|
||||||
|
channel.sendResponse(new XContentRestResponse(request, OK, builder));
|
||||||
|
} catch (IOException e) {
|
||||||
|
onFailure(e);
|
||||||
|
}
|
||||||
|
}
|
||||||
|
|
||||||
|
@Override public void onFailure(Throwable e) {
|
||||||
|
try {
|
||||||
|
Throwable t = unwrapCause(e);
|
||||||
|
if (t instanceof IndexMissingException) {
|
||||||
|
XContentBuilder builder = RestXContentBuilder.restContentBuilder(request);
|
||||||
|
channel.sendResponse(new XContentRestResponse(request, BAD_REQUEST, builder.startObject().field("error", t.getMessage()).endObject()));
|
||||||
|
} else {
|
||||||
|
channel.sendResponse(new XContentThrowableRestResponse(request, e));
|
||||||
|
}
|
||||||
|
} catch (IOException e1) {
|
||||||
|
logger.error("Failed to send failure response", e1);
|
||||||
|
}
|
||||||
|
}
|
||||||
|
});
|
||||||
|
}
|
||||||
|
|
||||||
|
static final class Fields {
|
||||||
|
static final XContentBuilderString OK = new XContentBuilderString("ok");
|
||||||
|
static final XContentBuilderString ACKNOWLEDGED = new XContentBuilderString("acknowledged");
|
||||||
|
}
|
||||||
|
}
|
|
@ -0,0 +1,87 @@
|
||||||
|
/*
|
||||||
|
* 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.rest.action.admin.indices.open;
|
||||||
|
|
||||||
|
import org.elasticsearch.action.ActionListener;
|
||||||
|
import org.elasticsearch.action.admin.indices.open.OpenIndexRequest;
|
||||||
|
import org.elasticsearch.action.admin.indices.open.OpenIndexResponse;
|
||||||
|
import org.elasticsearch.client.Client;
|
||||||
|
import org.elasticsearch.common.inject.Inject;
|
||||||
|
import org.elasticsearch.common.settings.Settings;
|
||||||
|
import org.elasticsearch.common.xcontent.XContentBuilder;
|
||||||
|
import org.elasticsearch.common.xcontent.XContentBuilderString;
|
||||||
|
import org.elasticsearch.indices.IndexMissingException;
|
||||||
|
import org.elasticsearch.rest.*;
|
||||||
|
import org.elasticsearch.rest.action.support.RestXContentBuilder;
|
||||||
|
|
||||||
|
import java.io.IOException;
|
||||||
|
|
||||||
|
import static org.elasticsearch.ExceptionsHelper.*;
|
||||||
|
import static org.elasticsearch.common.unit.TimeValue.*;
|
||||||
|
import static org.elasticsearch.rest.RestResponse.Status.*;
|
||||||
|
|
||||||
|
/**
|
||||||
|
* @author kimchy (shay.banon)
|
||||||
|
*/
|
||||||
|
public class RestOpenIndexAction extends BaseRestHandler {
|
||||||
|
|
||||||
|
@Inject public RestOpenIndexAction(Settings settings, Client client, RestController controller) {
|
||||||
|
super(settings, client);
|
||||||
|
controller.registerHandler(RestRequest.Method.POST, "/{index}/_open", this);
|
||||||
|
}
|
||||||
|
|
||||||
|
@Override public void handleRequest(final RestRequest request, final RestChannel channel) {
|
||||||
|
OpenIndexRequest openIndexRequest = new OpenIndexRequest(request.param("index"));
|
||||||
|
openIndexRequest.timeout(request.paramAsTime("timeout", timeValueSeconds(10)));
|
||||||
|
client.admin().indices().open(openIndexRequest, new ActionListener<OpenIndexResponse>() {
|
||||||
|
@Override public void onResponse(OpenIndexResponse response) {
|
||||||
|
try {
|
||||||
|
XContentBuilder builder = RestXContentBuilder.restContentBuilder(request);
|
||||||
|
builder.startObject()
|
||||||
|
.field(Fields.OK, true)
|
||||||
|
.field(Fields.ACKNOWLEDGED, response.acknowledged())
|
||||||
|
.endObject();
|
||||||
|
channel.sendResponse(new XContentRestResponse(request, OK, builder));
|
||||||
|
} catch (IOException e) {
|
||||||
|
onFailure(e);
|
||||||
|
}
|
||||||
|
}
|
||||||
|
|
||||||
|
@Override public void onFailure(Throwable e) {
|
||||||
|
try {
|
||||||
|
Throwable t = unwrapCause(e);
|
||||||
|
if (t instanceof IndexMissingException) {
|
||||||
|
XContentBuilder builder = RestXContentBuilder.restContentBuilder(request);
|
||||||
|
channel.sendResponse(new XContentRestResponse(request, BAD_REQUEST, builder.startObject().field("error", t.getMessage()).endObject()));
|
||||||
|
} else {
|
||||||
|
channel.sendResponse(new XContentThrowableRestResponse(request, e));
|
||||||
|
}
|
||||||
|
} catch (IOException e1) {
|
||||||
|
logger.error("Failed to send failure response", e1);
|
||||||
|
}
|
||||||
|
}
|
||||||
|
});
|
||||||
|
}
|
||||||
|
|
||||||
|
static final class Fields {
|
||||||
|
static final XContentBuilderString OK = new XContentBuilderString("ok");
|
||||||
|
static final XContentBuilderString ACKNOWLEDGED = new XContentBuilderString("acknowledged");
|
||||||
|
}
|
||||||
|
}
|
|
@ -0,0 +1,141 @@
|
||||||
|
/*
|
||||||
|
* 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.test.integration.gateway.local;
|
||||||
|
|
||||||
|
import org.elasticsearch.action.admin.cluster.health.ClusterHealthResponse;
|
||||||
|
import org.elasticsearch.action.admin.cluster.state.ClusterStateResponse;
|
||||||
|
import org.elasticsearch.action.get.GetResponse;
|
||||||
|
import org.elasticsearch.cluster.block.ClusterBlockException;
|
||||||
|
import org.elasticsearch.cluster.metadata.IndexMetaData;
|
||||||
|
import org.elasticsearch.cluster.routing.ShardRoutingState;
|
||||||
|
import org.elasticsearch.common.logging.ESLogger;
|
||||||
|
import org.elasticsearch.common.logging.Loggers;
|
||||||
|
import org.elasticsearch.gateway.Gateway;
|
||||||
|
import org.elasticsearch.node.internal.InternalNode;
|
||||||
|
import org.elasticsearch.test.integration.AbstractNodesTests;
|
||||||
|
import org.testng.annotations.AfterMethod;
|
||||||
|
import org.testng.annotations.Test;
|
||||||
|
|
||||||
|
import static org.elasticsearch.common.settings.ImmutableSettings.*;
|
||||||
|
import static org.hamcrest.MatcherAssert.*;
|
||||||
|
import static org.hamcrest.Matchers.*;
|
||||||
|
|
||||||
|
/**
|
||||||
|
* @author kimchy (shay.banon)
|
||||||
|
*/
|
||||||
|
public class LocalGatewayIndexStateTests extends AbstractNodesTests {
|
||||||
|
|
||||||
|
private final ESLogger logger = Loggers.getLogger(LocalGatewayIndexStateTests.class);
|
||||||
|
|
||||||
|
@AfterMethod public void cleanAndCloseNodes() throws Exception {
|
||||||
|
for (int i = 0; i < 10; i++) {
|
||||||
|
if (node("node" + i) != null) {
|
||||||
|
node("node" + i).stop();
|
||||||
|
// since we store (by default) the index snapshot under the gateway, resetting it will reset the index data as well
|
||||||
|
((InternalNode) node("node" + i)).injector().getInstance(Gateway.class).reset();
|
||||||
|
}
|
||||||
|
}
|
||||||
|
closeAllNodes();
|
||||||
|
}
|
||||||
|
|
||||||
|
@Test public void testSimpleOpenClose() throws Exception {
|
||||||
|
logger.info("--> cleaning nodes");
|
||||||
|
buildNode("node1", settingsBuilder().put("gateway.type", "local").build());
|
||||||
|
buildNode("node2", settingsBuilder().put("gateway.type", "local").build());
|
||||||
|
cleanAndCloseNodes();
|
||||||
|
|
||||||
|
logger.info("--> starting 2 nodes");
|
||||||
|
startNode("node1", settingsBuilder().put("gateway.type", "local").put("index.number_of_shards", 2).put("index.number_of_replicas", 1).build());
|
||||||
|
startNode("node2", settingsBuilder().put("gateway.type", "local").put("index.number_of_shards", 2).put("index.number_of_replicas", 1).build());
|
||||||
|
|
||||||
|
logger.info("--> creating test index");
|
||||||
|
client("node1").admin().indices().prepareCreate("test").execute().actionGet();
|
||||||
|
|
||||||
|
logger.info("--> waiting for green status");
|
||||||
|
ClusterHealthResponse health = client("node1").admin().cluster().prepareHealth().setWaitForGreenStatus().setWaitForNodes("2").execute().actionGet();
|
||||||
|
assertThat(health.timedOut(), equalTo(false));
|
||||||
|
|
||||||
|
ClusterStateResponse stateResponse = client("node1").admin().cluster().prepareState().execute().actionGet();
|
||||||
|
assertThat(stateResponse.state().metaData().index("test").state(), equalTo(IndexMetaData.State.OPEN));
|
||||||
|
assertThat(stateResponse.state().routingTable().index("test").shards().size(), equalTo(2));
|
||||||
|
assertThat(stateResponse.state().routingTable().index("test").shardsWithState(ShardRoutingState.STARTED).size(), equalTo(4));
|
||||||
|
|
||||||
|
logger.info("--> indexing a simple document");
|
||||||
|
client("node1").prepareIndex("test", "type1", "1").setSource("field1", "value1").execute().actionGet();
|
||||||
|
|
||||||
|
logger.info("--> closing test index...");
|
||||||
|
client("node1").admin().indices().prepareClose("test").execute().actionGet();
|
||||||
|
|
||||||
|
stateResponse = client("node1").admin().cluster().prepareState().execute().actionGet();
|
||||||
|
assertThat(stateResponse.state().metaData().index("test").state(), equalTo(IndexMetaData.State.CLOSE));
|
||||||
|
assertThat(stateResponse.state().routingTable().index("test"), nullValue());
|
||||||
|
|
||||||
|
logger.info("--> trying to index into a closed index ...");
|
||||||
|
try {
|
||||||
|
client("node1").prepareIndex("test", "type1", "1").setSource("field1", "value1").execute().actionGet();
|
||||||
|
assert false;
|
||||||
|
} catch (ClusterBlockException e) {
|
||||||
|
// all is well
|
||||||
|
}
|
||||||
|
|
||||||
|
logger.info("--> closing nodes...");
|
||||||
|
closeNode("node2");
|
||||||
|
closeNode("node1");
|
||||||
|
|
||||||
|
logger.info("--> starting nodes again...");
|
||||||
|
startNode("node1", settingsBuilder().put("gateway.type", "local").build());
|
||||||
|
startNode("node2", settingsBuilder().put("gateway.type", "local").build());
|
||||||
|
|
||||||
|
logger.info("--> waiting for green status");
|
||||||
|
health = client("node1").admin().cluster().prepareHealth().setWaitForGreenStatus().setWaitForNodes("2").execute().actionGet();
|
||||||
|
assertThat(health.timedOut(), equalTo(false));
|
||||||
|
|
||||||
|
stateResponse = client("node1").admin().cluster().prepareState().execute().actionGet();
|
||||||
|
assertThat(stateResponse.state().metaData().index("test").state(), equalTo(IndexMetaData.State.CLOSE));
|
||||||
|
assertThat(stateResponse.state().routingTable().index("test"), nullValue());
|
||||||
|
|
||||||
|
logger.info("--> trying to index into a closed index ...");
|
||||||
|
try {
|
||||||
|
client("node1").prepareIndex("test", "type1", "1").setSource("field1", "value1").execute().actionGet();
|
||||||
|
assert false;
|
||||||
|
} catch (ClusterBlockException e) {
|
||||||
|
// all is well
|
||||||
|
}
|
||||||
|
|
||||||
|
logger.info("--> opening index...");
|
||||||
|
client("node1").admin().indices().prepareOpen("test").execute().actionGet();
|
||||||
|
|
||||||
|
logger.info("--> waiting for green status");
|
||||||
|
health = client("node1").admin().cluster().prepareHealth().setWaitForGreenStatus().setWaitForNodes("2").execute().actionGet();
|
||||||
|
assertThat(health.timedOut(), equalTo(false));
|
||||||
|
|
||||||
|
stateResponse = client("node1").admin().cluster().prepareState().execute().actionGet();
|
||||||
|
assertThat(stateResponse.state().metaData().index("test").state(), equalTo(IndexMetaData.State.OPEN));
|
||||||
|
assertThat(stateResponse.state().routingTable().index("test").shards().size(), equalTo(2));
|
||||||
|
assertThat(stateResponse.state().routingTable().index("test").shardsWithState(ShardRoutingState.STARTED).size(), equalTo(4));
|
||||||
|
|
||||||
|
logger.info("--> trying to get the indexed document on the first round (before close and shutdown)");
|
||||||
|
GetResponse getResponse = client("node1").prepareGet("test", "type1", "1").execute().actionGet();
|
||||||
|
assertThat(getResponse.exists(), equalTo(true));
|
||||||
|
|
||||||
|
logger.info("--> indexing a simple document");
|
||||||
|
client("node1").prepareIndex("test", "type1", "2").setSource("field1", "value1").execute().actionGet();
|
||||||
|
}
|
||||||
|
}
|
|
@ -0,0 +1,101 @@
|
||||||
|
/*
|
||||||
|
* 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.test.integration.indices.state;
|
||||||
|
|
||||||
|
import org.elasticsearch.action.admin.cluster.health.ClusterHealthResponse;
|
||||||
|
import org.elasticsearch.action.admin.cluster.state.ClusterStateResponse;
|
||||||
|
import org.elasticsearch.action.admin.indices.status.IndicesStatusResponse;
|
||||||
|
import org.elasticsearch.cluster.block.ClusterBlockException;
|
||||||
|
import org.elasticsearch.cluster.metadata.IndexMetaData;
|
||||||
|
import org.elasticsearch.cluster.routing.ShardRoutingState;
|
||||||
|
import org.elasticsearch.common.logging.ESLogger;
|
||||||
|
import org.elasticsearch.common.logging.Loggers;
|
||||||
|
import org.elasticsearch.test.integration.AbstractNodesTests;
|
||||||
|
import org.testng.annotations.AfterMethod;
|
||||||
|
import org.testng.annotations.Test;
|
||||||
|
|
||||||
|
import static org.hamcrest.MatcherAssert.*;
|
||||||
|
import static org.hamcrest.Matchers.*;
|
||||||
|
|
||||||
|
/**
|
||||||
|
* @author kimchy (shay.banon)
|
||||||
|
*/
|
||||||
|
public class SimpleIndexStateTests extends AbstractNodesTests {
|
||||||
|
|
||||||
|
private final ESLogger logger = Loggers.getLogger(SimpleIndexStateTests.class);
|
||||||
|
|
||||||
|
@AfterMethod public void closeNodes() {
|
||||||
|
closeAllNodes();
|
||||||
|
}
|
||||||
|
|
||||||
|
@Test public void testSimpleOpenClose() {
|
||||||
|
logger.info("--> starting two nodes....");
|
||||||
|
startNode("node1");
|
||||||
|
startNode("node2");
|
||||||
|
|
||||||
|
logger.info("--> creating test index");
|
||||||
|
client("node1").admin().indices().prepareCreate("test").execute().actionGet();
|
||||||
|
|
||||||
|
logger.info("--> waiting for green status");
|
||||||
|
ClusterHealthResponse health = client("node1").admin().cluster().prepareHealth().setWaitForGreenStatus().setWaitForNodes("2").execute().actionGet();
|
||||||
|
assertThat(health.timedOut(), equalTo(false));
|
||||||
|
|
||||||
|
ClusterStateResponse stateResponse = client("node1").admin().cluster().prepareState().execute().actionGet();
|
||||||
|
assertThat(stateResponse.state().metaData().index("test").state(), equalTo(IndexMetaData.State.OPEN));
|
||||||
|
assertThat(stateResponse.state().routingTable().index("test").shards().size(), equalTo(5));
|
||||||
|
assertThat(stateResponse.state().routingTable().index("test").shardsWithState(ShardRoutingState.STARTED).size(), equalTo(10));
|
||||||
|
|
||||||
|
logger.info("--> indexing a simple document");
|
||||||
|
client("node1").prepareIndex("test", "type1", "1").setSource("field1", "value1").execute().actionGet();
|
||||||
|
|
||||||
|
logger.info("--> closing test index...");
|
||||||
|
client("node1").admin().indices().prepareClose("test").execute().actionGet();
|
||||||
|
|
||||||
|
stateResponse = client("node1").admin().cluster().prepareState().execute().actionGet();
|
||||||
|
assertThat(stateResponse.state().metaData().index("test").state(), equalTo(IndexMetaData.State.CLOSE));
|
||||||
|
assertThat(stateResponse.state().routingTable().index("test"), nullValue());
|
||||||
|
|
||||||
|
logger.info("--> testing indices status api...");
|
||||||
|
IndicesStatusResponse indicesStatusResponse = client("node1").admin().indices().prepareStatus().execute().actionGet();
|
||||||
|
|
||||||
|
logger.info("--> trying to index into a closed index ...");
|
||||||
|
try {
|
||||||
|
client("node1").prepareIndex("test", "type1", "1").setSource("field1", "value1").execute().actionGet();
|
||||||
|
assert false;
|
||||||
|
} catch (ClusterBlockException e) {
|
||||||
|
// all is well
|
||||||
|
}
|
||||||
|
|
||||||
|
logger.info("--> opening index...");
|
||||||
|
client("node1").admin().indices().prepareOpen("test").execute().actionGet();
|
||||||
|
|
||||||
|
logger.info("--> waiting for green status");
|
||||||
|
health = client("node1").admin().cluster().prepareHealth().setWaitForGreenStatus().setWaitForNodes("2").execute().actionGet();
|
||||||
|
assertThat(health.timedOut(), equalTo(false));
|
||||||
|
|
||||||
|
stateResponse = client("node1").admin().cluster().prepareState().execute().actionGet();
|
||||||
|
assertThat(stateResponse.state().metaData().index("test").state(), equalTo(IndexMetaData.State.OPEN));
|
||||||
|
assertThat(stateResponse.state().routingTable().index("test").shards().size(), equalTo(5));
|
||||||
|
assertThat(stateResponse.state().routingTable().index("test").shardsWithState(ShardRoutingState.STARTED).size(), equalTo(10));
|
||||||
|
|
||||||
|
logger.info("--> indexing a simple document");
|
||||||
|
client("node1").prepareIndex("test", "type1", "1").setSource("field1", "value1").execute().actionGet();
|
||||||
|
}
|
||||||
|
}
|
Loading…
Reference in New Issue