Revert "SOLR-11072: Implement trigger for searchRate event type."

This reverts commit 8ef83bf
This commit is contained in:
Shalin Shekhar Mangar 2017-09-18 09:26:49 -07:00
parent bc25e846bc
commit 341a730ad5
15 changed files with 228 additions and 277 deletions

View File

@ -69,7 +69,7 @@ Upgrade Notes
* SOLR-11254: the abstract DocTransformer class now has an abstract score-less transform method variant.
* SOLR-11283: all Stream Evaluators in solrj.io.eval have been refactored to have a simplier and more
robust structure. This simplifies and condenses the code required to implement a new Evaluator and
robust structure. This simplifies and condenses the code required to implement a new Evaluator and
makes it much easier for evaluators to handle differing data types (primitives, objects, arrays,
lists, and so forth). (Dennis Gove)
@ -98,8 +98,6 @@ New Features
* SOLR-11244: Query DSL for Solr (Cao Manh Dat)
* SOLR-11072: Implement trigger for searchRate event type. (ab)
* SOLR-11317: JSON Facet API: min/max aggregations on numeric fields are now typed better so int/long
fields return an appropriate integral type rather than a double. (yonik)

View File

@ -20,20 +20,15 @@ package org.apache.solr.cloud.autoscaling;
import java.io.IOException;
import java.lang.invoke.MethodHandles;
import java.util.ArrayList;
import java.util.HashSet;
import java.util.List;
import java.util.Map;
import java.util.Set;
import org.apache.solr.client.solrj.SolrRequest;
import org.apache.solr.client.solrj.cloud.autoscaling.AutoScalingConfig;
import org.apache.solr.client.solrj.cloud.autoscaling.NoneSuggester;
import org.apache.solr.client.solrj.cloud.autoscaling.Policy;
import org.apache.solr.client.solrj.cloud.autoscaling.ReplicaInfo;
import org.apache.solr.client.solrj.impl.CloudSolrClient;
import org.apache.solr.client.solrj.impl.SolrClientDataProvider;
import org.apache.solr.common.cloud.ZkStateReader;
import org.apache.solr.common.params.AutoScalingParams;
import org.apache.solr.common.params.CollectionParams;
import org.apache.solr.core.CoreContainer;
import org.apache.zookeeper.KeeperException;
@ -107,35 +102,6 @@ public class ComputePlanAction extends TriggerActionBase {
.hint(Policy.Suggester.Hint.SRC_NODE, event.getProperty(TriggerEvent.NODE_NAMES));
log.debug("Created suggester with srcNode: {}", event.getProperty(TriggerEvent.NODE_NAMES));
break;
case SEARCHRATE:
Map<String, Map<String, Double>> hotShards = (Map<String, Map<String, Double>>)event.getProperty(AutoScalingParams.SHARD);
Map<String, Double> hotCollections = (Map<String, Double>)event.getProperty(AutoScalingParams.COLLECTION);
List<ReplicaInfo> hotReplicas = (List<ReplicaInfo>)event.getProperty(AutoScalingParams.REPLICA);
Map<String, Double> hotNodes = (Map<String, Double>)event.getProperty(AutoScalingParams.NODE);
if (hotShards.isEmpty() && hotCollections.isEmpty() && hotReplicas.isEmpty()) {
// node -> MOVEREPLICA
if (hotNodes.isEmpty()) {
log.warn("Neither hot replicas / collection nor nodes are reported in event: " + event);
return NoneSuggester.INSTANCE;
}
suggester = session.getSuggester(CollectionParams.CollectionAction.MOVEREPLICA);
for (String node : hotNodes.keySet()) {
suggester = suggester.hint(Policy.Suggester.Hint.SRC_NODE, node);
}
} else {
// collection || shard || replica -> ADDREPLICA
suggester = session.getSuggester(CollectionParams.CollectionAction.ADDREPLICA);
Set<String> collections = new HashSet<>();
// XXX improve this when AddReplicaSuggester supports coll_shard hint
hotReplicas.forEach(r -> collections.add(r.getCollection()));
hotShards.forEach((coll, shards) -> collections.add(coll));
hotCollections.forEach((coll, rate) -> collections.add(coll));
for (String coll : collections) {
suggester = suggester.hint(Policy.Suggester.Hint.COLL, coll);
}
}
break;
default:
throw new UnsupportedOperationException("No support for events other than nodeAdded and nodeLost, received: " + event.getEventType());
}

View File

@ -17,17 +17,23 @@
package org.apache.solr.cloud.autoscaling;
import java.io.IOException;
import java.lang.invoke.MethodHandles;
import java.util.ArrayList;
import java.util.Collection;
import java.util.Collections;
import java.util.HashMap;
import java.util.HashSet;
import java.util.Iterator;
import java.util.List;
import java.util.Locale;
import java.util.Map;
import java.util.Objects;
import java.util.Set;
import java.util.concurrent.TimeUnit;
import java.util.concurrent.atomic.AtomicReference;
import org.apache.lucene.util.IOUtils;
import org.apache.solr.client.solrj.cloud.autoscaling.TriggerEventType;
import org.apache.solr.common.SolrException;
import org.apache.solr.common.cloud.ZkStateReader;
@ -43,26 +49,57 @@ import org.slf4j.LoggerFactory;
public class NodeAddedTrigger extends TriggerBase {
private static final Logger log = LoggerFactory.getLogger(MethodHandles.lookup().lookupClass());
private final String name;
private final Map<String, Object> properties;
private final CoreContainer container;
private final List<TriggerAction> actions;
private final AtomicReference<AutoScaling.TriggerEventProcessor> processorRef;
private final boolean enabled;
private final int waitForSecond;
private final TriggerEventType eventType;
private final TimeSource timeSource;
private boolean isClosed = false;
private Set<String> lastLiveNodes;
private Map<String, Long> nodeNameVsTimeAdded = new HashMap<>();
public NodeAddedTrigger(String name, Map<String, Object> properties,
CoreContainer container) {
super(TriggerEventType.NODEADDED, name, properties, container.getResourceLoader(), container.getZkController().getZkClient());
super(container.getZkController().getZkClient());
this.name = name;
this.properties = properties;
this.container = container;
this.timeSource = TimeSource.CURRENT_TIME;
this.processorRef = new AtomicReference<>();
List<Map<String, String>> o = (List<Map<String, String>>) properties.get("actions");
if (o != null && !o.isEmpty()) {
actions = new ArrayList<>(3);
for (Map<String, String> map : o) {
TriggerAction action = container.getResourceLoader().newInstance(map.get("class"), TriggerAction.class);
actions.add(action);
}
} else {
actions = Collections.emptyList();
}
lastLiveNodes = new HashSet<>(container.getZkController().getZkStateReader().getClusterState().getLiveNodes());
log.debug("Initial livenodes: {}", lastLiveNodes);
this.enabled = Boolean.parseBoolean(String.valueOf(properties.getOrDefault("enabled", "true")));
this.waitForSecond = ((Long) properties.getOrDefault("waitFor", -1L)).intValue();
this.eventType = TriggerEventType.valueOf(properties.get("event").toString().toUpperCase(Locale.ROOT));
log.debug("NodeAddedTrigger {} instantiated with properties: {}", name, properties);
}
@Override
public void init() {
super.init();
List<Map<String, String>> o = (List<Map<String, String>>) properties.get("actions");
if (o != null && !o.isEmpty()) {
for (int i = 0; i < o.size(); i++) {
Map<String, String> map = o.get(i);
actions.get(i).init(map);
}
}
// pick up added nodes for which marker paths were created
try {
List<String> added = container.getZkController().getZkClient().getChildren(ZkStateReader.SOLR_AUTOSCALING_NODE_ADDED_PATH, null, true);
@ -82,6 +119,69 @@ public class NodeAddedTrigger extends TriggerBase {
}
@Override
public void setProcessor(AutoScaling.TriggerEventProcessor processor) {
processorRef.set(processor);
}
@Override
public AutoScaling.TriggerEventProcessor getProcessor() {
return processorRef.get();
}
@Override
public String getName() {
return name;
}
@Override
public TriggerEventType getEventType() {
return eventType;
}
@Override
public boolean isEnabled() {
return enabled;
}
@Override
public int getWaitForSecond() {
return waitForSecond;
}
@Override
public Map<String, Object> getProperties() {
return properties;
}
@Override
public List<TriggerAction> getActions() {
return actions;
}
@Override
public boolean equals(Object obj) {
if (obj instanceof NodeAddedTrigger) {
NodeAddedTrigger that = (NodeAddedTrigger) obj;
return this.name.equals(that.name)
&& this.properties.equals(that.properties);
}
return false;
}
@Override
public int hashCode() {
return Objects.hash(name, properties);
}
@Override
public void close() throws IOException {
synchronized (this) {
isClosed = true;
IOUtils.closeWhileHandlingException(actions);
}
}
@Override
public void restoreState(AutoScaling.Trigger old) {
assert old.isClosed();
@ -198,6 +298,13 @@ public class NodeAddedTrigger extends TriggerBase {
}
@Override
public boolean isClosed() {
synchronized (this) {
return isClosed;
}
}
public static class NodeAddedEvent extends TriggerEvent {
public NodeAddedEvent(TriggerEventType eventType, String source, List<Long> times, List<String> nodeNames) {

View File

@ -17,17 +17,23 @@
package org.apache.solr.cloud.autoscaling;
import java.io.IOException;
import java.lang.invoke.MethodHandles;
import java.util.ArrayList;
import java.util.Collection;
import java.util.Collections;
import java.util.HashMap;
import java.util.HashSet;
import java.util.Iterator;
import java.util.List;
import java.util.Locale;
import java.util.Map;
import java.util.Objects;
import java.util.Set;
import java.util.concurrent.TimeUnit;
import java.util.concurrent.atomic.AtomicReference;
import org.apache.lucene.util.IOUtils;
import org.apache.solr.client.solrj.cloud.autoscaling.TriggerEventType;
import org.apache.solr.common.SolrException;
import org.apache.solr.common.cloud.ZkStateReader;
@ -43,7 +49,14 @@ import org.slf4j.LoggerFactory;
public class NodeLostTrigger extends TriggerBase {
private static final Logger log = LoggerFactory.getLogger(MethodHandles.lookup().lookupClass());
private final String name;
private final Map<String, Object> properties;
private final CoreContainer container;
private final List<TriggerAction> actions;
private final AtomicReference<AutoScaling.TriggerEventProcessor> processorRef;
private final boolean enabled;
private final int waitForSecond;
private final TriggerEventType eventType;
private final TimeSource timeSource;
private boolean isClosed = false;
@ -54,11 +67,27 @@ public class NodeLostTrigger extends TriggerBase {
public NodeLostTrigger(String name, Map<String, Object> properties,
CoreContainer container) {
super(TriggerEventType.NODELOST, name, properties, container.getResourceLoader(), container.getZkController().getZkClient());
super(container.getZkController().getZkClient());
this.name = name;
this.properties = properties;
this.container = container;
this.timeSource = TimeSource.CURRENT_TIME;
this.processorRef = new AtomicReference<>();
List<Map<String, String>> o = (List<Map<String, String>>) properties.get("actions");
if (o != null && !o.isEmpty()) {
actions = new ArrayList<>(3);
for (Map<String, String> map : o) {
TriggerAction action = container.getResourceLoader().newInstance(map.get("class"), TriggerAction.class);
actions.add(action);
}
} else {
actions = Collections.emptyList();
}
lastLiveNodes = new HashSet<>(container.getZkController().getZkStateReader().getClusterState().getLiveNodes());
log.debug("Initial livenodes: {}", lastLiveNodes);
this.enabled = Boolean.parseBoolean(String.valueOf(properties.getOrDefault("enabled", "true")));
this.waitForSecond = ((Long) properties.getOrDefault("waitFor", -1L)).intValue();
this.eventType = TriggerEventType.valueOf(properties.get("event").toString().toUpperCase(Locale.ROOT));
}
@Override
@ -88,6 +117,69 @@ public class NodeLostTrigger extends TriggerBase {
}
}
@Override
public void setProcessor(AutoScaling.TriggerEventProcessor processor) {
processorRef.set(processor);
}
@Override
public AutoScaling.TriggerEventProcessor getProcessor() {
return processorRef.get();
}
@Override
public String getName() {
return name;
}
@Override
public TriggerEventType getEventType() {
return eventType;
}
@Override
public boolean isEnabled() {
return enabled;
}
@Override
public int getWaitForSecond() {
return waitForSecond;
}
@Override
public Map<String, Object> getProperties() {
return properties;
}
@Override
public List<TriggerAction> getActions() {
return actions;
}
@Override
public boolean equals(Object obj) {
if (obj instanceof NodeLostTrigger) {
NodeLostTrigger that = (NodeLostTrigger) obj;
return this.name.equals(that.name)
&& this.properties.equals(that.properties);
}
return false;
}
@Override
public int hashCode() {
return Objects.hash(name, properties);
}
@Override
public void close() throws IOException {
synchronized (this) {
isClosed = true;
IOUtils.closeWhileHandlingException(actions);
}
}
@Override
public void restoreState(AutoScaling.Trigger old) {
assert old.isClosed();
@ -204,6 +296,13 @@ public class NodeLostTrigger extends TriggerBase {
}
}
@Override
public boolean isClosed() {
synchronized (this) {
return isClosed;
}
}
public static class NodeLostEvent extends TriggerEvent {
public NodeLostEvent(TriggerEventType eventType, String source, List<Long> times, List<String> nodeNames) {

View File

@ -16,25 +16,14 @@
*/
package org.apache.solr.cloud.autoscaling;
import java.io.IOException;
import java.lang.invoke.MethodHandles;
import java.util.ArrayList;
import java.util.Arrays;
import java.util.Collections;
import java.util.HashMap;
import java.util.List;
import java.util.Locale;
import java.util.Map;
import java.util.Objects;
import java.util.TreeMap;
import java.util.concurrent.atomic.AtomicReference;
import org.apache.lucene.util.IOUtils;
import org.apache.solr.client.solrj.cloud.autoscaling.TriggerEventType;
import org.apache.solr.common.cloud.SolrZkClient;
import org.apache.solr.common.cloud.ZkStateReader;
import org.apache.solr.common.util.Utils;
import org.apache.solr.core.SolrResourceLoader;
import org.apache.zookeeper.CreateMode;
import org.apache.zookeeper.KeeperException;
import org.apache.zookeeper.data.Stat;
@ -48,129 +37,19 @@ import org.slf4j.LoggerFactory;
public abstract class TriggerBase implements AutoScaling.Trigger {
private static final Logger LOG = LoggerFactory.getLogger(MethodHandles.lookup().lookupClass());
protected final String name;
protected final Map<String, Object> properties = new HashMap<>();
protected final TriggerEventType eventType;
protected final int waitForSecond;
protected SolrZkClient zkClient;
protected Map<String,Object> lastState;
protected final List<TriggerAction> actions;
protected final AtomicReference<AutoScaling.TriggerEventProcessor> processorRef = new AtomicReference<>();
protected final boolean enabled;
protected boolean isClosed;
protected TriggerBase(TriggerEventType eventType, String name, Map<String, Object> properties, SolrResourceLoader loader, SolrZkClient zkClient) {
this.eventType = eventType;
this.name = name;
protected TriggerBase(SolrZkClient zkClient) {
this.zkClient = zkClient;
if (properties != null) {
this.properties.putAll(properties);
}
this.enabled = Boolean.parseBoolean(String.valueOf(this.properties.getOrDefault("enabled", "true")));
this.waitForSecond = ((Number) this.properties.getOrDefault("waitFor", -1L)).intValue();
List<Map<String, String>> o = (List<Map<String, String>>) properties.get("actions");
if (o != null && !o.isEmpty()) {
actions = new ArrayList<>(3);
for (Map<String, String> map : o) {
TriggerAction action = loader.newInstance(map.get("class"), TriggerAction.class);
actions.add(action);
}
} else {
actions = Collections.emptyList();
}
try {
if (!zkClient.exists(ZkStateReader.SOLR_AUTOSCALING_TRIGGER_STATE_PATH, true)) {
zkClient.makePath(ZkStateReader.SOLR_AUTOSCALING_TRIGGER_STATE_PATH, false, true);
}
zkClient.makePath(ZkStateReader.SOLR_AUTOSCALING_TRIGGER_STATE_PATH, false, true);
} catch (KeeperException | InterruptedException e) {
LOG.warn("Exception checking ZK path " + ZkStateReader.SOLR_AUTOSCALING_TRIGGER_STATE_PATH, e);
}
}
@Override
public void init() {
List<Map<String, String>> o = (List<Map<String, String>>) properties.get("actions");
if (o != null && !o.isEmpty()) {
for (int i = 0; i < o.size(); i++) {
Map<String, String> map = o.get(i);
actions.get(i).init(map);
}
}
}
@Override
public void setProcessor(AutoScaling.TriggerEventProcessor processor) {
processorRef.set(processor);
}
@Override
public AutoScaling.TriggerEventProcessor getProcessor() {
return processorRef.get();
}
@Override
public String getName() {
return name;
}
@Override
public TriggerEventType getEventType() {
return eventType;
}
@Override
public boolean isEnabled() {
return enabled;
}
@Override
public int getWaitForSecond() {
return waitForSecond;
}
@Override
public Map<String, Object> getProperties() {
return properties;
}
@Override
public List<TriggerAction> getActions() {
return actions;
}
@Override
public boolean equals(Object obj) {
if (obj == null) {
return false;
}
if (obj.getClass().equals(this.getClass())) {
TriggerBase that = (TriggerBase) obj;
return this.name.equals(that.name)
&& this.properties.equals(that.properties);
}
return false;
}
@Override
public int hashCode() {
return Objects.hash(name, properties);
}
@Override
public void close() throws IOException {
synchronized (this) {
isClosed = true;
IOUtils.closeWhileHandlingException(actions);
}
}
@Override
public boolean isClosed() {
synchronized (this) {
return isClosed;
}
}
/**
* Prepare and return internal state of this trigger in a format suitable for persisting in ZK.
* @return map of internal state properties. Note: values must be supported by {@link Utils#toJSON(Object)}.

View File

@ -22,7 +22,6 @@ import java.util.Collections;
import java.util.EnumSet;
import java.util.HashSet;
import java.util.List;
import java.util.Map;
import java.util.Set;
import java.util.regex.Pattern;
import java.util.stream.Collectors;
@ -140,18 +139,14 @@ public class MetricsHandler extends RequestHandlerBase implements PermissionName
continue;
}
MetricUtils.PropertyFilter propertyFilter = MetricUtils.PropertyFilter.ALL;
boolean simple = false;
if (propertyName != null) {
propertyFilter = (name) -> name.equals(propertyName);
simple = true;
// use escaped versions
key = parts[0] + ":" + parts[1];
}
MetricUtils.convertMetric(key, m, propertyFilter, false, true, true, false, ":", (k, v) -> {
if ((v instanceof Map) && propertyName != null) {
((Map)v).forEach((k1, v1) -> result.add(k + ":" + k1, v1));
} else {
result.add(k, v);
}
});
MetricUtils.convertMetric(key, m, propertyFilter, false, true, true, simple, ":", (k, v) -> result.add(k, v));
}
rsp.getValues().add("metrics", result);
if (errors.size() > 0) {

View File

@ -277,15 +277,6 @@ public class MetricsHandlerTest extends SolrTestCaseJ4 {
val = values.findRecursive("metrics", key3);
assertNotNull(val);
String key4 = "solr.core.collection1:QUERY./select.requestTimes:1minRate";
resp = new SolrQueryResponse();
handler.handleRequestBody(req(CommonParams.QT, "/admin/metrics", CommonParams.WT, "json",
MetricsHandler.KEY_PARAM, key4), resp);
values = resp.getValues();
val = values.findRecursive("metrics", key4);
assertNotNull(val);
assertTrue(val instanceof Number);
// test errors
// invalid keys

View File

@ -23,8 +23,6 @@ import java.util.Collection;
import java.util.List;
import java.util.Map;
import org.apache.solr.common.cloud.ClusterState;
public interface ClusterDataProvider extends Closeable {
/**Get the value of each tag for a given node
*
@ -44,8 +42,6 @@ public interface ClusterDataProvider extends Closeable {
Collection<String> getNodes();
ClusterState getClusterState();
/**Get the collection-specific policy
*/
String getPolicyNameByCollection(String coll);

View File

@ -19,9 +19,7 @@ package org.apache.solr.client.solrj.cloud.autoscaling;
import org.apache.solr.client.solrj.SolrRequest;
public class NoneSuggester extends Policy.Suggester {
public static final NoneSuggester INSTANCE = new NoneSuggester();
public class NoneSuggester extends Policy.Suggester{
@Override
SolrRequest init() {
return null;

View File

@ -203,7 +203,7 @@ public class Policy implements MapWriter {
}
Session(ClusterDataProvider dataProvider) {
this.nodes = new ArrayList<>(dataProvider.getClusterState().getLiveNodes());
this.nodes = new ArrayList<>(dataProvider.getNodes());
this.dataProvider = dataProvider;
for (String node : nodes) {
collections.addAll(dataProvider.getReplicaInfo(node, Collections.emptyList()).keySet());

View File

@ -30,7 +30,6 @@ import java.util.concurrent.atomic.AtomicLong;
import org.apache.solr.client.solrj.SolrRequest;
import org.apache.solr.client.solrj.cloud.autoscaling.Policy.Suggester.Hint;
import org.apache.solr.common.SolrException;
import org.apache.solr.common.cloud.ClusterState;
import org.apache.solr.common.cloud.Replica;
import org.apache.solr.common.cloud.ReplicaPosition;
import org.apache.solr.common.util.Pair;
@ -67,11 +66,6 @@ public class PolicyHelper {
return delegate.getNodes();
}
@Override
public ClusterState getClusterState() {
return delegate.getClusterState();
}
@Override
public String getPolicyNameByCollection(String coll) {
return policyMapping.get() != null && policyMapping.get().containsKey(coll) ?

View File

@ -36,7 +36,6 @@ public class ReplicaInfo implements MapWriter {
this.collection = coll;
this.shard = shard;
this.type = type;
this.core = (String)vals.get("core");
}
@Override
@ -55,10 +54,6 @@ public class ReplicaInfo implements MapWriter {
return core;
}
public String getName() {
return name;
}
public String getCollection() {
return collection;
}
@ -66,23 +61,4 @@ public class ReplicaInfo implements MapWriter {
public String getShard() {
return shard;
}
public Map<String, Object> getVariables() {
return variables;
}
public Object getVariable(String name) {
return variables != null ? variables.get(name) : null;
}
@Override
public String toString() {
return "ReplicaInfo{" +
"name='" + name + '\'' +
", collection='" + collection + '\'' +
", shard='" + shard + '\'' +
", type=" + type +
", variables=" + variables +
'}';
}
}

View File

@ -64,14 +64,15 @@ public class SolrClientDataProvider implements ClusterDataProvider, MapWriter {
private final CloudSolrClient solrClient;
private final Map<String, Map<String, Map<String, List<ReplicaInfo>>>> data = new HashMap<>();
private ClusterState clusterState;
private Set<String> liveNodes;
private Map<String, Object> snitchSession = new HashMap<>();
private Map<String, Map> nodeVsTags = new HashMap<>();
public SolrClientDataProvider(CloudSolrClient solrClient) {
this.solrClient = solrClient;
ZkStateReader zkStateReader = solrClient.getZkStateReader();
clusterState = zkStateReader.getClusterState();
ClusterState clusterState = zkStateReader.getClusterState();
this.liveNodes = clusterState.getLiveNodes();
Map<String, ClusterState.CollectionRef> all = clusterState.getCollectionStates();
all.forEach((collName, ref) -> {
DocCollection coll = ref.get();
@ -80,21 +81,11 @@ public class SolrClientDataProvider implements ClusterDataProvider, MapWriter {
Map<String, Map<String, List<ReplicaInfo>>> nodeData = data.computeIfAbsent(replica.getNodeName(), k -> new HashMap<>());
Map<String, List<ReplicaInfo>> collData = nodeData.computeIfAbsent(collName, k -> new HashMap<>());
List<ReplicaInfo> replicas = collData.computeIfAbsent(shard, k -> new ArrayList<>());
replicas.add(new ReplicaInfo(replica.getName(), collName, shard, replica.getType(), replica.getProperties()));
replicas.add(new ReplicaInfo(replica.getName(), collName, shard, replica.getType(), new HashMap<>()));
});
});
}
@Override
public Collection<String> getNodes() {
return clusterState.getLiveNodes();
}
@Override
public ClusterState getClusterState() {
return clusterState;
}
@Override
public String getPolicyNameByCollection(String coll) {
ClusterState.CollectionRef state = solrClient.getClusterStateProvider().getState(coll);
@ -115,9 +106,14 @@ public class SolrClientDataProvider implements ClusterDataProvider, MapWriter {
return data.computeIfAbsent(node, s -> Collections.emptyMap());//todo fill other details
}
@Override
public Collection<String> getNodes() {
return liveNodes;
}
@Override
public void writeMap(EntryWriter ew) throws IOException {
ew.put("clusterState", clusterState);
ew.put("liveNodes", liveNodes);
ew.put("replicaInfo", Utils.getDeepCopy(data, 5));
ew.put("nodeValues", nodeVsTags);

View File

@ -37,12 +37,6 @@ public interface AutoScalingParams {
String BEFORE_ACTION = "beforeAction";
String AFTER_ACTION = "afterAction";
String TIMEOUT = "timeout";
String COLLECTION = "collection";
String SHARD = "shard";
String REPLICA = "replica";
String NODE = "node";
String HANDLER = "handler";
String RATE = "rate";
String REMOVE_LISTENERS = "removeListeners";
String ZK_VERSION = "zkVersion";

View File

@ -34,7 +34,6 @@ import org.apache.solr.client.solrj.SolrRequest;
import org.apache.solr.client.solrj.cloud.autoscaling.Clause.Violation;
import org.apache.solr.client.solrj.cloud.autoscaling.Policy.Suggester.Hint;
import org.apache.solr.client.solrj.request.CollectionAdminRequest;
import org.apache.solr.common.cloud.ClusterState;
import org.apache.solr.common.cloud.Replica;
import org.apache.solr.common.cloud.ReplicaPosition;
import org.apache.solr.common.cloud.ZkStateReader;
@ -439,11 +438,6 @@ public class TestPolicy extends SolrTestCaseJ4 {
return (Collection<String>) m.get("liveNodes");
}
@Override
public ClusterState getClusterState() {
throw new UnsupportedOperationException("getClusterState");
}
@Override
public String getPolicyNameByCollection(String coll) {
return null;
@ -974,11 +968,6 @@ public class TestPolicy extends SolrTestCaseJ4 {
return clusterDataProvider.getNodes();
}
@Override
public ClusterState getClusterState() {
return clusterDataProvider.getClusterState();
}
@Override
public String getPolicyNameByCollection(String coll) {
return null;
@ -1054,11 +1043,6 @@ public class TestPolicy extends SolrTestCaseJ4 {
return replicaInfoMap.keySet();
}
@Override
public ClusterState getClusterState() {
throw new UnsupportedOperationException("getClusterState");
}
@Override
public String getPolicyNameByCollection(String coll) {
return null;
@ -1117,11 +1101,6 @@ public class TestPolicy extends SolrTestCaseJ4 {
return clusterDataProvider.getNodes();
}
@Override
public ClusterState getClusterState() {
return clusterDataProvider.getClusterState();
}
@Override
public String getPolicyNameByCollection(String coll) {
return "p1";
@ -1150,11 +1129,6 @@ public class TestPolicy extends SolrTestCaseJ4 {
return nodeValues.keySet();
}
@Override
public ClusterState getClusterState() {
throw new UnsupportedOperationException("getClusterState");
}
@Override
public String getPolicyNameByCollection(String coll) {
return null;
@ -1201,12 +1175,6 @@ public class TestPolicy extends SolrTestCaseJ4 {
public Collection<String> getNodes() {
return Arrays.asList( "127.0.0.1:50097_solr", "127.0.0.1:50096_solr");
}
@Override
public ClusterState getClusterState() {
throw new UnsupportedOperationException("getClusterState");
}
};
List<ReplicaPosition> locations = PolicyHelper.getReplicaLocations(
"newColl", new AutoScalingConfig((Map<String, Object>)Utils.fromJSONString(autoScaleJson)),
@ -1264,12 +1232,6 @@ public class TestPolicy extends SolrTestCaseJ4 {
public Collection<String> getNodes() {
return Arrays.asList("node1", "node2", "node3", "node4");
}
@Override
public ClusterState getClusterState() {
throw new UnsupportedOperationException("getClusterState");
}
};
List<ReplicaPosition> locations = PolicyHelper.getReplicaLocations(
"newColl", new AutoScalingConfig((Map<String, Object>) Utils.fromJSONString(autoScaleJson)),