River name re-use not possible between node shutdowns, closes #1921.

This effectively happens because we don't flush before deleting a mapping, which causes it to still reply the changes happening for it on the transaction log, meaning the mapping are still around on the index level.
This commit is contained in:
Shay Banon 2012-06-23 20:26:27 +02:00
parent 6fb836c25e
commit ded5b773da
1 changed files with 24 additions and 8 deletions

View File

@ -21,6 +21,8 @@ package org.elasticsearch.action.admin.indices.mapping.delete;
import org.elasticsearch.ElasticSearchException;
import org.elasticsearch.action.ActionListener;
import org.elasticsearch.action.admin.indices.flush.FlushResponse;
import org.elasticsearch.action.admin.indices.flush.TransportFlushAction;
import org.elasticsearch.action.admin.indices.refresh.RefreshResponse;
import org.elasticsearch.action.admin.indices.refresh.TransportRefreshAction;
import org.elasticsearch.action.deletebyquery.DeleteByQueryResponse;
@ -49,6 +51,8 @@ public class TransportDeleteMappingAction extends TransportMasterNodeOperationAc
private final MetaDataMappingService metaDataMappingService;
private final TransportFlushAction flushAction;
private final TransportDeleteByQueryAction deleteByQueryAction;
private final TransportRefreshAction refreshAction;
@ -56,11 +60,12 @@ public class TransportDeleteMappingAction extends TransportMasterNodeOperationAc
@Inject
public TransportDeleteMappingAction(Settings settings, TransportService transportService, ClusterService clusterService,
ThreadPool threadPool, MetaDataMappingService metaDataMappingService,
TransportDeleteByQueryAction deleteByQueryAction, TransportRefreshAction refreshAction) {
TransportDeleteByQueryAction deleteByQueryAction, TransportRefreshAction refreshAction, TransportFlushAction flushAction) {
super(settings, transportService, clusterService, threadPool);
this.metaDataMappingService = metaDataMappingService;
this.deleteByQueryAction = deleteByQueryAction;
this.refreshAction = refreshAction;
this.flushAction = flushAction;
}
@Override
@ -100,6 +105,9 @@ public class TransportDeleteMappingAction extends TransportMasterNodeOperationAc
final AtomicReference<Throwable> failureRef = new AtomicReference<Throwable>();
final CountDownLatch latch = new CountDownLatch(1);
flushAction.execute(Requests.flushRequest(request.indices()), new ActionListener<FlushResponse>() {
@Override
public void onResponse(FlushResponse flushResponse) {
deleteByQueryAction.execute(Requests.deleteByQueryRequest(request.indices()).query(QueryBuilders.filteredQuery(QueryBuilders.matchAllQuery(), FilterBuilders.typeFilter(request.type()))), new ActionListener<DeleteByQueryResponse>() {
@Override
public void onResponse(DeleteByQueryResponse deleteByQueryResponse) {
@ -124,6 +132,14 @@ public class TransportDeleteMappingAction extends TransportMasterNodeOperationAc
latch.countDown();
}
});
}
@Override
public void onFailure(Throwable e) {
failureRef.set(e);
latch.countDown();
}
});
try {
latch.await();