001/**
002 * Licensed to the Apache Software Foundation (ASF) under one
003 * or more contributor license agreements.  See the NOTICE file
004 * distributed with this work for additional information
005 * regarding copyright ownership.  The ASF licenses this file
006 * to you under the Apache License, Version 2.0 (the
007 * "License"); you may not use this file except in compliance
008 * with the License.  You may obtain a copy of the License at
009 *
010 *     http://www.apache.org/licenses/LICENSE-2.0
011 *
012 * Unless required by applicable law or agreed to in writing, software
013 * distributed under the License is distributed on an "AS IS" BASIS,
014 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
015 * See the License for the specific language governing permissions and
016 * limitations under the License.
017 */
018package org.apache.hadoop.hbase.rsgroup;
019
020import java.io.IOException;
021import java.util.ArrayList;
022import java.util.Collections;
023import java.util.HashMap;
024import java.util.HashSet;
025import java.util.Iterator;
026import java.util.LinkedList;
027import java.util.List;
028import java.util.Map;
029import java.util.Set;
030import org.apache.commons.lang3.StringUtils;
031import org.apache.hadoop.hbase.NamespaceDescriptor;
032import org.apache.hadoop.hbase.ServerName;
033import org.apache.hadoop.hbase.TableName;
034import org.apache.hadoop.hbase.client.RegionInfo;
035import org.apache.hadoop.hbase.constraint.ConstraintException;
036import org.apache.hadoop.hbase.master.HMaster;
037import org.apache.hadoop.hbase.master.LoadBalancer;
038import org.apache.hadoop.hbase.master.MasterServices;
039import org.apache.hadoop.hbase.master.RegionPlan;
040import org.apache.hadoop.hbase.master.RegionState;
041import org.apache.hadoop.hbase.master.ServerManager;
042import org.apache.hadoop.hbase.master.assignment.AssignmentManager;
043import org.apache.hadoop.hbase.master.assignment.RegionStateNode;
044import org.apache.hadoop.hbase.net.Address;
045import org.apache.yetus.audience.InterfaceAudience;
046import org.slf4j.Logger;
047import org.slf4j.LoggerFactory;
048
049import org.apache.hbase.thirdparty.com.google.common.collect.Maps;
050
051/**
052 * Service to support Region Server Grouping (HBase-6721).
053 */
054@InterfaceAudience.Private
055public class RSGroupAdminServer implements RSGroupAdmin {
056  private static final Logger LOG = LoggerFactory.getLogger(RSGroupAdminServer.class);
057  public static final String KEEP_ONE_SERVER_IN_DEFAULT_ERROR_MESSAGE = "should keep at least " +
058          "one server in 'default' RSGroup.";
059
060  private MasterServices master;
061  private final RSGroupInfoManager rsGroupInfoManager;
062
063  public RSGroupAdminServer(MasterServices master, RSGroupInfoManager rsGroupInfoManager) {
064    this.master = master;
065    this.rsGroupInfoManager = rsGroupInfoManager;
066  }
067
068  @Override
069  public RSGroupInfo getRSGroupInfo(String groupName) throws IOException {
070    return rsGroupInfoManager.getRSGroup(groupName);
071  }
072
073  @Override
074  public RSGroupInfo getRSGroupInfoOfTable(TableName tableName) throws IOException {
075    // We are reading across two Maps in the below with out synchronizing across
076    // them; should be safe most of the time.
077    String groupName = rsGroupInfoManager.getRSGroupOfTable(tableName);
078    return groupName == null? null: rsGroupInfoManager.getRSGroup(groupName);
079  }
080
081  private void checkOnlineServersOnly(Set<Address> servers) throws ConstraintException {
082    // This uglyness is because we only have Address, not ServerName.
083    // Online servers are keyed by ServerName.
084    Set<Address> onlineServers = new HashSet<>();
085    for(ServerName server: master.getServerManager().getOnlineServers().keySet()) {
086      onlineServers.add(server.getAddress());
087    }
088    for (Address address: servers) {
089      if (!onlineServers.contains(address)) {
090        throw new ConstraintException(
091            "Server " + address + " is not an online server in 'default' RSGroup.");
092      }
093    }
094  }
095
096  /**
097   * Check passed name. Fail if nulls or if corresponding RSGroupInfo not found.
098   * @return The RSGroupInfo named <code>name</code>
099   */
100  private RSGroupInfo getAndCheckRSGroupInfo(String name) throws IOException {
101    if (StringUtils.isEmpty(name)) {
102      throw new ConstraintException("RSGroup cannot be null.");
103    }
104    RSGroupInfo rsGroupInfo = getRSGroupInfo(name);
105    if (rsGroupInfo == null) {
106      throw new ConstraintException("RSGroup does not exist: " + name);
107    }
108    return rsGroupInfo;
109  }
110
111  /**
112   * @return List of Regions associated with this <code>server</code>.
113   */
114  private List<RegionInfo> getRegions(final Address server) {
115    LinkedList<RegionInfo> regions = new LinkedList<>();
116    for (Map.Entry<RegionInfo, ServerName> el :
117        master.getAssignmentManager().getRegionStates().getRegionAssignments().entrySet()) {
118      if (el.getValue() == null) {
119        continue;
120      }
121
122      if (el.getValue().getAddress().equals(server)) {
123        addRegion(regions, el.getKey());
124      }
125    }
126    for (RegionStateNode state : master.getAssignmentManager().getRegionsInTransition()) {
127      if (state.getRegionLocation() != null &&
128          state.getRegionLocation().getAddress().equals(server)) {
129        addRegion(regions, state.getRegionInfo());
130      }
131    }
132    return regions;
133  }
134
135  private void addRegion(final LinkedList<RegionInfo> regions, RegionInfo hri) {
136    // If meta, move it last otherwise other unassigns fail because meta is not
137    // online for them to update state in. This is dodgy. Needs to be made more
138    // robust. See TODO below.
139    if (hri.isMetaRegion()) {
140      regions.addLast(hri);
141    } else {
142      regions.addFirst(hri);
143    }
144  }
145
146  /**
147   * Check servers and tables.
148   *
149   * @param servers servers to move
150   * @param tables tables to move
151   * @param targetGroupName target group name
152   * @throws IOException if nulls or if servers and tables not belong to the same group
153   */
154  private void checkServersAndTables(Set<Address> servers, Set<TableName> tables,
155                                     String targetGroupName) throws IOException {
156    // Presume first server's source group. Later ensure all servers are from this group.
157    Address firstServer = servers.iterator().next();
158    RSGroupInfo tmpSrcGrp = rsGroupInfoManager.getRSGroupOfServer(firstServer);
159    if (tmpSrcGrp == null) {
160      // Be careful. This exception message is tested for in TestRSGroupsBase...
161      throw new ConstraintException("Source RSGroup for server " + firstServer
162              + " does not exist.");
163    }
164    RSGroupInfo srcGrp = new RSGroupInfo(tmpSrcGrp);
165
166    // Only move online servers
167    checkOnlineServersOnly(servers);
168
169    // Ensure all servers are of same rsgroup.
170    for (Address server: servers) {
171      String tmpGroup = rsGroupInfoManager.getRSGroupOfServer(server).getName();
172      if (!tmpGroup.equals(srcGrp.getName())) {
173        throw new ConstraintException("Move server request should only come from one source " +
174                "RSGroup. Expecting only " + srcGrp.getName() + " but contains " + tmpGroup);
175      }
176    }
177
178    // Ensure all tables and servers are of same rsgroup.
179    for (TableName table : tables) {
180      String tmpGroup = rsGroupInfoManager.getRSGroupOfTable(table);
181      if (!tmpGroup.equals(srcGrp.getName())) {
182        throw new ConstraintException("Move table request should only come from one source " +
183                "RSGroup. Expecting only " + srcGrp.getName() + " but contains " + tmpGroup);
184      }
185    }
186
187    if (srcGrp.getServers().size() <= servers.size() && srcGrp.getTables().size() > tables.size()) {
188      throw new ConstraintException("Cannot leave a RSGroup " + srcGrp.getName() +
189              " that contains tables without servers to host them.");
190    }
191  }
192
193  /**
194   * Move every region from servers which are currently located on these servers,
195   * but should not be located there.
196   *
197   * @param servers the servers that will move to new group
198   * @param targetGroupName the target group name
199   * @throws IOException if moving the server and tables fail
200   */
201  private void moveServerRegionsFromGroup(Set<Address> servers, String targetGroupName)
202      throws IOException {
203    boolean hasRegionsToMove;
204    int retry = 0;
205    RSGroupInfo targetGrp = getRSGroupInfo(targetGroupName);
206    Set<Address> allSevers = new HashSet<>(servers);
207    do {
208      hasRegionsToMove = false;
209      for (Iterator<Address> iter = allSevers.iterator(); iter.hasNext();) {
210        Address rs = iter.next();
211        // Get regions that are associated with this server and filter regions by group tables.
212        for (RegionInfo region : getRegions(rs)) {
213          if (!targetGrp.containsTable(region.getTable())) {
214            LOG.info("Moving server region {}, which do not belong to RSGroup {}",
215                region.getShortNameToLog(), targetGroupName);
216            try {
217              this.master.getAssignmentManager().move(region);
218            }catch (IOException ioe){
219              LOG.error("Move region {} from group failed, will retry, current retry time is {}",
220                  region.getShortNameToLog(), retry, ioe);
221            }
222            if (master.getAssignmentManager().getRegionStates().
223                getRegionState(region).isFailedOpen()) {
224              continue;
225            }
226            hasRegionsToMove = true;
227          }
228        }
229
230        if (!hasRegionsToMove) {
231          LOG.info("Server {} has no more regions to move for RSGroup", rs.getHostname());
232          iter.remove();
233        }
234      }
235
236      retry++;
237      try {
238        rsGroupInfoManager.wait(1000);
239      } catch (InterruptedException e) {
240        LOG.warn("Sleep interrupted", e);
241        Thread.currentThread().interrupt();
242      }
243    } while (hasRegionsToMove && retry <= 50);
244  }
245
246  /**
247   * Moves regions of tables which are not on target group servers.
248   *
249   * @param tables the tables that will move to new group
250   * @param targetGroupName the target group name
251   * @throws IOException if moving the region fails
252   */
253  private void moveTableRegionsToGroup(Set<TableName> tables, String targetGroupName)
254      throws IOException {
255    boolean hasRegionsToMove;
256    int retry = 0;
257    RSGroupInfo targetGrp = getRSGroupInfo(targetGroupName);
258    Set<TableName> allTables = new HashSet<>(tables);
259    do {
260      hasRegionsToMove = false;
261      for (Iterator<TableName> iter = allTables.iterator(); iter.hasNext(); ) {
262        TableName table = iter.next();
263        if (master.getAssignmentManager().isTableDisabled(table)) {
264          LOG.debug("Skipping move regions because the table {} is disabled", table);
265          continue;
266        }
267        LOG.info("Moving region(s) for table {} to RSGroup {}", table, targetGroupName);
268        for (RegionInfo region : master.getAssignmentManager().getRegionStates()
269            .getRegionsOfTable(table)) {
270          ServerName sn =
271              master.getAssignmentManager().getRegionStates().getRegionServerOfRegion(region);
272          if (!targetGrp.containsServer(sn.getAddress())) {
273            LOG.info("Moving region {} to RSGroup {}", region.getShortNameToLog(), targetGroupName);
274            try {
275              master.getAssignmentManager().move(region);
276            }catch (IOException ioe){
277              LOG.error("Move region {} to group failed, will retry, current retry time is {}",
278                  region.getShortNameToLog(), retry, ioe);
279            }
280            hasRegionsToMove = true;
281          }
282        }
283
284        if (!hasRegionsToMove) {
285          LOG.info("Table {} has no more regions to move for RSGroup {}", table.getNameAsString(),
286            targetGroupName);
287          iter.remove();
288        }
289      }
290
291      retry++;
292      try {
293        rsGroupInfoManager.wait(1000);
294      } catch (InterruptedException e) {
295        LOG.warn("Sleep interrupted", e);
296        Thread.currentThread().interrupt();
297      }
298    } while (hasRegionsToMove && retry <= 50);
299  }
300
301  @edu.umd.cs.findbugs.annotations.SuppressWarnings(
302      value="RCN_REDUNDANT_NULLCHECK_WOULD_HAVE_BEEN_A_NPE",
303      justification="Ignoring complaint because don't know what it is complaining about")
304  @Override
305  public void moveServers(Set<Address> servers, String targetGroupName) throws IOException {
306    if (servers == null) {
307      throw new ConstraintException("The list of servers to move cannot be null.");
308    }
309    if (servers.isEmpty()) {
310      // For some reason this difference between null servers and isEmpty is important distinction.
311      // TODO. Why? Stuff breaks if I equate them.
312      return;
313    }
314    //check target group
315    getAndCheckRSGroupInfo(targetGroupName);
316
317    // Hold a lock on the manager instance while moving servers to prevent
318    // another writer changing our state while we are working.
319    synchronized (rsGroupInfoManager) {
320      // Presume first server's source group. Later ensure all servers are from this group.
321      Address firstServer = servers.iterator().next();
322      RSGroupInfo srcGrp = rsGroupInfoManager.getRSGroupOfServer(firstServer);
323      if (srcGrp == null) {
324        // Be careful. This exception message is tested for in TestRSGroupsBase...
325        throw new ConstraintException("Source RSGroup for server " + firstServer
326            + " does not exist.");
327      }
328      // Only move online servers (when moving from 'default') or servers from other
329      // groups. This prevents bogus servers from entering groups
330      if (RSGroupInfo.DEFAULT_GROUP.equals(srcGrp.getName())) {
331        if (srcGrp.getServers().size() <= servers.size()) {
332          throw new ConstraintException(KEEP_ONE_SERVER_IN_DEFAULT_ERROR_MESSAGE);
333        }
334        checkOnlineServersOnly(servers);
335      }
336      // Ensure all servers are of same rsgroup.
337      for (Address server: servers) {
338        String tmpGroup = rsGroupInfoManager.getRSGroupOfServer(server).getName();
339        if (!tmpGroup.equals(srcGrp.getName())) {
340          throw new ConstraintException("Move server request should only come from one source " +
341              "RSGroup. Expecting only " + srcGrp.getName() + " but contains " + tmpGroup);
342        }
343      }
344      if (srcGrp.getServers().size() <= servers.size() && srcGrp.getTables().size() > 0) {
345        throw new ConstraintException("Cannot leave a RSGroup " + srcGrp.getName() +
346            " that contains tables without servers to host them.");
347      }
348
349      // MovedServers may be < passed in 'servers'.
350      Set<Address> movedServers = rsGroupInfoManager.moveServers(servers, srcGrp.getName(),
351          targetGroupName);
352      moveServerRegionsFromGroup(movedServers, targetGroupName);
353      LOG.info("Move servers done: {} => {}", srcGrp.getName(), targetGroupName);
354    }
355  }
356
357  @Override
358  public void moveTables(Set<TableName> tables, String targetGroup) throws IOException {
359    if (tables == null) {
360      throw new ConstraintException("The list of tables cannot be null.");
361    }
362    if (tables.size() < 1) {
363      LOG.debug("moveTables() passed an empty set. Ignoring.");
364      return;
365    }
366
367    // Hold a lock on the manager instance while moving servers to prevent
368    // another writer changing our state while we are working.
369    synchronized (rsGroupInfoManager) {
370      if(targetGroup != null) {
371        RSGroupInfo destGroup = rsGroupInfoManager.getRSGroup(targetGroup);
372        if(destGroup == null) {
373          throw new ConstraintException("Target " + targetGroup + " RSGroup does not exist.");
374        }
375        if(destGroup.getServers().size() < 1) {
376          throw new ConstraintException("Target RSGroup must have at least one server.");
377        }
378      }
379      rsGroupInfoManager.moveTables(tables, targetGroup);
380
381      // targetGroup is null when a table is being deleted. In this case no further
382      // action is required.
383      if (targetGroup != null) {
384        moveTableRegionsToGroup(tables, targetGroup);
385      }
386    }
387  }
388
389  @Override
390  public void addRSGroup(String name) throws IOException {
391    rsGroupInfoManager.addRSGroup(new RSGroupInfo(name));
392  }
393
394  @Override
395  public void removeRSGroup(String name) throws IOException {
396    // Hold a lock on the manager instance while moving servers to prevent
397    // another writer changing our state while we are working.
398    synchronized (rsGroupInfoManager) {
399      RSGroupInfo rsGroupInfo = rsGroupInfoManager.getRSGroup(name);
400      if (rsGroupInfo == null) {
401        throw new ConstraintException("RSGroup " + name + " does not exist");
402      }
403      int tableCount = rsGroupInfo.getTables().size();
404      if (tableCount > 0) {
405        throw new ConstraintException("RSGroup " + name + " has " + tableCount +
406            " tables; you must remove these tables from the rsgroup before " +
407            "the rsgroup can be removed.");
408      }
409      int serverCount = rsGroupInfo.getServers().size();
410      if (serverCount > 0) {
411        throw new ConstraintException("RSGroup " + name + " has " + serverCount +
412            " servers; you must remove these servers from the RSGroup before" +
413            "the RSGroup can be removed.");
414      }
415      for (NamespaceDescriptor ns : master.getClusterSchema().getNamespaces()) {
416        String nsGroup = ns.getConfigurationValue(RSGroupInfo.NAMESPACE_DESC_PROP_GROUP);
417        if (nsGroup != null && nsGroup.equals(name)) {
418          throw new ConstraintException(
419            "RSGroup " + name + " is referenced by namespace: " + ns.getName());
420        }
421      }
422      rsGroupInfoManager.removeRSGroup(name);
423    }
424  }
425
426  @Override
427  public boolean balanceRSGroup(String groupName) throws IOException {
428    ServerManager serverManager = master.getServerManager();
429    LoadBalancer balancer = master.getLoadBalancer();
430
431    synchronized (balancer) {
432      // If balance not true, don't run balancer.
433      if (!((HMaster) master).isBalancerOn()) {
434        return false;
435      }
436
437      if (getRSGroupInfo(groupName) == null) {
438        throw new ConstraintException("RSGroup does not exist: "+groupName);
439      }
440      // Only allow one balance run at at time.
441      Map<String, RegionState> groupRIT = rsGroupGetRegionsInTransition(groupName);
442      if (groupRIT.size() > 0) {
443        LOG.debug("Not running balancer because {} region(s) in transition: {}", groupRIT.size(),
444            StringUtils.abbreviate(
445              master.getAssignmentManager().getRegionStates().getRegionsInTransition().toString(),
446              256));
447        return false;
448      }
449      if (serverManager.areDeadServersInProgress()) {
450        LOG.debug("Not running balancer because processing dead regionserver(s): {}",
451            serverManager.getDeadServers());
452        return false;
453      }
454
455      //We balance per group instead of per table
456      Map<TableName, Map<ServerName, List<RegionInfo>>> assignmentsByTable =
457        getRSGroupAssignmentsByTable(groupName);
458      List<RegionPlan> plans = balancer.balanceCluster(assignmentsByTable);
459      boolean balancerRan = !plans.isEmpty();
460      if (balancerRan) {
461        LOG.info("RSGroup balance {} starting with plan count: {}", groupName, plans.size());
462        master.executeRegionPlansWithThrottling(plans);
463        LOG.info("RSGroup balance " + groupName + " completed");
464      }
465      return balancerRan;
466    }
467  }
468
469  @Override
470  public List<RSGroupInfo> listRSGroups() throws IOException {
471    return rsGroupInfoManager.listRSGroups();
472  }
473
474  @Override
475  public RSGroupInfo getRSGroupOfServer(Address hostPort) throws IOException {
476    return rsGroupInfoManager.getRSGroupOfServer(hostPort);
477  }
478
479  @Override
480  public void moveServersAndTables(Set<Address> servers, Set<TableName> tables, String targetGroup)
481      throws IOException {
482    if (servers == null || servers.isEmpty()) {
483      throw new ConstraintException("The list of servers to move cannot be null or empty.");
484    }
485    if (tables == null || tables.isEmpty()) {
486      throw new ConstraintException("The list of tables to move cannot be null or empty.");
487    }
488
489    //check target group
490    getAndCheckRSGroupInfo(targetGroup);
491
492    // Hold a lock on the manager instance while moving servers and tables to prevent
493    // another writer changing our state while we are working.
494    synchronized (rsGroupInfoManager) {
495      //check servers and tables status
496      checkServersAndTables(servers, tables, targetGroup);
497
498      //Move servers and tables to a new group.
499      String srcGroup = getRSGroupOfServer(servers.iterator().next()).getName();
500      rsGroupInfoManager.moveServersAndTables(servers, tables, srcGroup, targetGroup);
501
502      //move regions on these servers which do not belong to group tables
503      moveServerRegionsFromGroup(servers, targetGroup);
504      //move regions of these tables which are not on group servers
505      moveTableRegionsToGroup(tables, targetGroup);
506    }
507    LOG.info("Move servers and tables done. Severs: {}, Tables: {} => {}", servers, tables,
508        targetGroup);
509  }
510
511  @Override
512  public void removeServers(Set<Address> servers) throws IOException {
513    {
514      if (servers == null || servers.isEmpty()) {
515        throw new ConstraintException("The set of servers to remove cannot be null or empty.");
516      }
517      // Hold a lock on the manager instance while moving servers to prevent
518      // another writer changing our state while we are working.
519      synchronized (rsGroupInfoManager) {
520        //check the set of servers
521        checkForDeadOrOnlineServers(servers);
522        rsGroupInfoManager.removeServers(servers);
523        LOG.info("Remove decommissioned servers {} from RSGroup done", servers);
524      }
525    }
526  }
527
528  @Override
529  public void renameRSGroup(String oldName, String newName) throws IOException {
530    synchronized (rsGroupInfoManager) {
531      rsGroupInfoManager.renameRSGroup(oldName, newName);
532    }
533  }
534
535  private Map<String, RegionState> rsGroupGetRegionsInTransition(String groupName)
536      throws IOException {
537    Map<String, RegionState> rit = Maps.newTreeMap();
538    AssignmentManager am = master.getAssignmentManager();
539    for(TableName tableName : getRSGroupInfo(groupName).getTables()) {
540      for(RegionInfo regionInfo: am.getRegionStates().getRegionsOfTable(tableName)) {
541        RegionState state = am.getRegionStates().getRegionTransitionState(regionInfo);
542        if(state != null) {
543          rit.put(regionInfo.getEncodedName(), state);
544        }
545      }
546    }
547    return rit;
548  }
549
550  private Map<TableName, Map<ServerName, List<RegionInfo>>>
551      getRSGroupAssignmentsByTable(String groupName) throws IOException {
552    Map<TableName, Map<ServerName, List<RegionInfo>>> result = Maps.newHashMap();
553    RSGroupInfo rsGroupInfo = getRSGroupInfo(groupName);
554    Map<TableName, Map<ServerName, List<RegionInfo>>> assignments = Maps.newHashMap();
555    for(Map.Entry<RegionInfo, ServerName> entry:
556        master.getAssignmentManager().getRegionStates().getRegionAssignments().entrySet()) {
557      TableName currTable = entry.getKey().getTable();
558      ServerName currServer = entry.getValue();
559      RegionInfo currRegion = entry.getKey();
560      if (rsGroupInfo.getTables().contains(currTable)) {
561        assignments.putIfAbsent(currTable, new HashMap<>());
562        assignments.get(currTable).putIfAbsent(currServer, new ArrayList<>());
563        assignments.get(currTable).get(currServer).add(currRegion);
564      }
565    }
566
567    Map<ServerName, List<RegionInfo>> serverMap = Maps.newHashMap();
568    for(ServerName serverName: master.getServerManager().getOnlineServers().keySet()) {
569      if(rsGroupInfo.getServers().contains(serverName.getAddress())) {
570        serverMap.put(serverName, Collections.emptyList());
571      }
572    }
573
574    // add all tables that are members of the group
575    for(TableName tableName : rsGroupInfo.getTables()) {
576      if(assignments.containsKey(tableName)) {
577        result.put(tableName, new HashMap<>());
578        result.get(tableName).putAll(serverMap);
579        result.get(tableName).putAll(assignments.get(tableName));
580        LOG.debug("Adding assignments for {}: {}", tableName, assignments.get(tableName));
581      }
582    }
583
584    return result;
585  }
586
587  /**
588   * Check if the set of servers are belong to dead servers list or online servers list.
589   * @param servers servers to remove
590   */
591  private void checkForDeadOrOnlineServers(Set<Address> servers) throws ConstraintException {
592    // This uglyness is because we only have Address, not ServerName.
593    Set<Address> onlineServers = new HashSet<>();
594    List<ServerName> drainingServers = master.getServerManager().getDrainingServersList();
595    for (ServerName server : master.getServerManager().getOnlineServers().keySet()) {
596      // Only online but not decommissioned servers are really online
597      if (!drainingServers.contains(server)) {
598        onlineServers.add(server.getAddress());
599      }
600    }
601
602    Set<Address> deadServers = new HashSet<>();
603    for(ServerName server: master.getServerManager().getDeadServers().copyServerNames()) {
604      deadServers.add(server.getAddress());
605    }
606
607    for (Address address: servers) {
608      if (onlineServers.contains(address)) {
609        throw new ConstraintException(
610            "Server " + address + " is an online server, not allowed to remove.");
611      }
612      if (deadServers.contains(address)) {
613        throw new ConstraintException(
614            "Server " + address + " is on the dead servers list,"
615                + " Maybe it will come back again, not allowed to remove.");
616      }
617    }
618  }
619}