You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@doris.apache.org by ka...@apache.org on 2020/11/03 12:23:59 UTC

[incubator-doris] branch master updated: [LoadBalance] make BeLoadRebalancer extends from base class Rebalancer (#4771)

This is an automated email from the ASF dual-hosted git repository.

kangkaisen pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/incubator-doris.git


The following commit(s) were added to refs/heads/master by this push:
     new 80d5f6e  [LoadBalance] make BeLoadRebalancer extends from base class Rebalancer (#4771)
80d5f6e is described below

commit 80d5f6e3d86364e81c0a5c2ffbc4549aacbd9d1d
Author: HuangWei <hu...@apache.org>
AuthorDate: Tue Nov 3 20:23:48 2020 +0800

    [LoadBalance] make BeLoadRebalancer extends from base class Rebalancer (#4771)
---
 .../{LoadBalancer.java => BeLoadRebalancer.java}   | 61 +++++---------
 .../java/org/apache/doris/clone/Rebalancer.java    | 96 ++++++++++++++++++++++
 .../org/apache/doris/clone/TabletScheduler.java    | 79 +++++++++++-------
 3 files changed, 165 insertions(+), 71 deletions(-)

diff --git a/fe/fe-core/src/main/java/org/apache/doris/clone/LoadBalancer.java b/fe/fe-core/src/main/java/org/apache/doris/clone/BeLoadRebalancer.java
similarity index 88%
rename from fe/fe-core/src/main/java/org/apache/doris/clone/LoadBalancer.java
rename to fe/fe-core/src/main/java/org/apache/doris/clone/BeLoadRebalancer.java
index f764f56..9e26978 100644
--- a/fe/fe-core/src/main/java/org/apache/doris/clone/LoadBalancer.java
+++ b/fe/fe-core/src/main/java/org/apache/doris/clone/BeLoadRebalancer.java
@@ -27,7 +27,6 @@ import org.apache.doris.clone.TabletSchedCtx.Priority;
 import org.apache.doris.clone.TabletScheduler.PathSlot;
 import org.apache.doris.system.Backend;
 import org.apache.doris.system.SystemInfoService;
-import org.apache.doris.task.AgentBatchTask;
 import org.apache.doris.thrift.TStorageMedium;
 
 import com.google.common.collect.Lists;
@@ -43,32 +42,16 @@ import java.util.Map;
 import java.util.Set;
 
 /*
- * LoadBalancer is responsible for 
+ * BeLoadRebalancer strategy:
  * 1. selecting alternative tablets from high load backends, and return them to tablet scheduler.
  * 2. given a tablet, find a backend to migration.
+ * 3. deleting the redundant replica in high load, so don't override getCachedSrcBackendId().
  */
-public class LoadBalancer {
-    private static final Logger LOG = LogManager.getLogger(LoadBalancer.class);
+public class BeLoadRebalancer extends Rebalancer {
+    private static final Logger LOG = LogManager.getLogger(BeLoadRebalancer.class);
 
-    private Map<String, ClusterLoadStatistic> statisticMap;
-    private TabletInvertedIndex invertedIndex;
-    private SystemInfoService infoService;
-
-    public LoadBalancer(Map<String, ClusterLoadStatistic> statisticMap) {
-        this.statisticMap = statisticMap;
-        this.invertedIndex = Catalog.getCurrentInvertedIndex();
-        this.infoService = Catalog.getCurrentSystemInfo();
-    }
-
-    public List<TabletSchedCtx> selectAlternativeTablets() {
-        List<TabletSchedCtx> alternativeTablets = Lists.newArrayList();
-        for (Map.Entry<String, ClusterLoadStatistic> entry : statisticMap.entrySet()) {
-            for (TStorageMedium medium : TStorageMedium.values()) {
-                alternativeTablets.addAll(selectAlternativeTabletsForCluster(entry.getKey(),
-                        entry.getValue(), medium));
-            }
-        }
-        return alternativeTablets;
+    public BeLoadRebalancer(SystemInfoService infoService, TabletInvertedIndex invertedIndex) {
+        super(infoService, invertedIndex);
     }
 
     /*
@@ -79,14 +62,15 @@ public class LoadBalancer {
      *         and whether it is benefit for balance (All these will be checked in tablet scheduler)
      *      2. Only select tablets from 'high' backends.
      *      3. Only select tablets from 'high' and 'mid' paths.
-     * 
+     *
      * Here we only select tablets from high load node, do not set its src or dest, all this will be set
      * when this tablet is being scheduled in tablet scheduler.
-     * 
+     *
      * NOTICE that we may select any available tablets here, ignore their state.
      * The state will be checked when being scheduled in tablet scheduler.
      */
-    private List<TabletSchedCtx> selectAlternativeTabletsForCluster(
+    @Override
+    protected List<TabletSchedCtx> selectAlternativeTabletsForCluster(
             String clusterName, ClusterLoadStatistic clusterStat, TStorageMedium medium) {
         List<TabletSchedCtx> alternativeTablets = Lists.newArrayList();
 
@@ -95,7 +79,7 @@ public class LoadBalancer {
         List<BackendLoadStatistic> midBEs = Lists.newArrayList();
         List<BackendLoadStatistic> highBEs = Lists.newArrayList();
         clusterStat.getBackendStatisticByClass(lowBEs, midBEs, highBEs, medium);
-        
+
         if (lowBEs.isEmpty() && highBEs.isEmpty()) {
             LOG.info("cluster is balance: {} with medium: {}. skip", clusterName, medium);
             return alternativeTablets;
@@ -108,7 +92,7 @@ public class LoadBalancer {
                     lowBEs.stream().mapToLong(BackendLoadStatistic::getBeId).toArray(), medium);
             return alternativeTablets;
         }
-        
+
         if (lowBEs.stream().noneMatch(BackendLoadStatistic::hasAvailDisk)) {
             LOG.info("all low load backends {} have no available disk with medium: {}. skip",
                     lowBEs.stream().mapToLong(BackendLoadStatistic::getBeId).toArray(), medium);
@@ -125,7 +109,8 @@ public class LoadBalancer {
         // choose tablets from high load backends.
         // BackendLoadStatistic is sorted by load score in ascend order,
         // so we need to traverse it from last to first
-        OUTER: for (int i = highBEs.size() - 1; i >= 0; i--) {
+        OUTER:
+        for (int i = highBEs.size() - 1; i >= 0; i--) {
             BackendLoadStatistic beStat = highBEs.get(i);
 
             // classify the paths.
@@ -135,7 +120,7 @@ public class LoadBalancer {
             beStat.getPathStatisticByClass(pathLow, pathMid, pathHigh, medium);
             // we only select tablets from available mid and high load path
             pathHigh.addAll(pathMid);
-            
+
             // get all tablets on this backend, and shuffle them for random selection
             List<Long> tabletIds = invertedIndex.getTabletIdsByBackendIdAndStorageMedium(beStat.getBeId(), medium);
             Collections.shuffle(tabletIds);
@@ -170,7 +155,7 @@ public class LoadBalancer {
                     if (tabletMeta == null) {
                         continue;
                     }
-                    
+
                     if (colocateTableIndex.isColocateTable(tabletMeta.getTableId())) {
                         continue;
                     }
@@ -200,7 +185,7 @@ public class LoadBalancer {
 
         LOG.info("select alternative tablets for cluster: {}, medium: {}, num: {}, detail: {}",
                 clusterName, medium, alternativeTablets.size(),
-                alternativeTablets.stream().mapToLong(t -> t.getTabletId()).toArray());
+                alternativeTablets.stream().mapToLong(TabletSchedCtx::getTabletId).toArray());
         return alternativeTablets;
     }
 
@@ -209,10 +194,9 @@ public class LoadBalancer {
      * 1. Check if this tablet has replica on high load backend. If not, the balance will be cancelled.
      *    If yes, select a replica as source replica.
      * 2. Select a low load backend as destination. And tablet should not has replica on this backend.
-     * 3. Create a clone task.
      */
-    public void createBalanceTask(TabletSchedCtx tabletCtx, Map<Long, PathSlot> backendsWorkingSlots,
-            AgentBatchTask batchTask) throws SchedException {
+    @Override
+    public void completeSchedCtx(TabletSchedCtx tabletCtx, Map<Long, PathSlot> backendsWorkingSlots) throws SchedException {
         ClusterLoadStatistic clusterStat = statisticMap.get(tabletCtx.getCluster());
         if (clusterStat == null) {
             throw new SchedException(Status.UNRECOVERABLE, "cluster does not exist");
@@ -274,7 +258,7 @@ public class LoadBalancer {
         // Select a low load backend as destination.
         boolean setDest = false;
         for (BackendLoadStatistic beStat : lowBe) {
-            if (beStat.isAvailable() && !replicas.stream().anyMatch(r -> r.getBackendId() == beStat.getBeId())) {
+            if (beStat.isAvailable() && replicas.stream().noneMatch(r -> r.getBackendId() == beStat.getBeId())) {
                 // check if on same host.
                 Backend lowBackend = infoService.getBackend(beStat.getBeId());
                 if (lowBackend == null) {
@@ -283,7 +267,7 @@ public class LoadBalancer {
                 if (hosts.contains(lowBackend.getHost())) {
                     continue;
                 }
-                
+
                 // no replica on this low load backend
                 // 1. check if this clone task can make the cluster more balance.
                 List<RootPathLoadStatistic> availPaths = Lists.newArrayList();
@@ -327,8 +311,5 @@ public class LoadBalancer {
         if (!setDest) {
             throw new SchedException(Status.SCHEDULE_FAILED, "unable to find low backend");
         }
-
-        // create clone task
-        batchTask.addTask(tabletCtx.createCloneReplicaAndTask());
     }
 }
diff --git a/fe/fe-core/src/main/java/org/apache/doris/clone/Rebalancer.java b/fe/fe-core/src/main/java/org/apache/doris/clone/Rebalancer.java
new file mode 100644
index 0000000..e0bf5f7
--- /dev/null
+++ b/fe/fe-core/src/main/java/org/apache/doris/clone/Rebalancer.java
@@ -0,0 +1,96 @@
+// Licensed to the Apache Software Foundation (ASF) under one
+// or more contributor license agreements.  See the NOTICE file
+// distributed with this work for additional information
+// regarding copyright ownership.  The ASF 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.apache.doris.clone;
+
+import org.apache.doris.catalog.TabletInvertedIndex;
+import org.apache.doris.clone.TabletScheduler.PathSlot;
+import org.apache.doris.system.SystemInfoService;
+import org.apache.doris.task.AgentBatchTask;
+import org.apache.doris.thrift.TStorageMedium;
+
+import com.google.common.collect.Lists;
+
+import java.util.HashMap;
+import java.util.List;
+import java.util.Map;
+
+
+/*
+ * Rebalancer is responsible for
+ * 1. selectAlternativeTablets: selecting alternative tablets by one rebalance strategy,
+ * and return them to tablet scheduler(maybe contains the concrete moves, or maybe not).
+ * 2. createBalanceTask: given a tablet, try to create a clone task for this tablet.
+ * 3. getToDeleteReplicaId: if the rebalance strategy wants to delete the specified replica,
+ * override this func to let TabletScheduler know in handling redundant replica.
+ * NOTICE:
+ * 1. Adding the selected tablets by TabletScheduler may not succeed at all. And the move may be failed in some other places.
+ * So the thing you need to know is, Rebalancer cannot know when the move is failed.
+ * 2. If you want to make sure the move is succeed, you can assume that it's succeed when getToDeleteReplicaId called.
+ */
+public abstract class Rebalancer {
+    // When Rebalancer init, the statisticMap is usually empty. So it's no need to be an arg.
+    // Only use updateLoadStatistic() to load stats.
+    protected Map<String, ClusterLoadStatistic> statisticMap = new HashMap<>();
+    protected TabletInvertedIndex invertedIndex;
+    protected SystemInfoService infoService;
+
+    public Rebalancer(SystemInfoService infoService, TabletInvertedIndex invertedIndex) {
+        this.infoService = infoService;
+        this.invertedIndex = invertedIndex;
+    }
+
+    public List<TabletSchedCtx> selectAlternativeTablets() {
+        List<TabletSchedCtx> alternativeTablets = Lists.newArrayList();
+        for (Map.Entry<String, ClusterLoadStatistic> entry : statisticMap.entrySet()) {
+            for (TStorageMedium medium : TStorageMedium.values()) {
+                alternativeTablets.addAll(selectAlternativeTabletsForCluster(entry.getKey(),
+                        entry.getValue(), medium));
+            }
+        }
+        return alternativeTablets;
+    }
+
+    // The return TabletSchedCtx should have the tablet id at least. {srcReplica, destBe} can be complete here or
+    // later(when createBalanceTask called).
+    protected abstract List<TabletSchedCtx> selectAlternativeTabletsForCluster(
+            String clusterName, ClusterLoadStatistic clusterStat, TStorageMedium medium);
+
+
+    public void createBalanceTask(TabletSchedCtx tabletCtx, Map<Long, PathSlot> backendsWorkingSlots,
+                                  AgentBatchTask batchTask) throws SchedException {
+        completeSchedCtx(tabletCtx, backendsWorkingSlots);
+        batchTask.addTask(tabletCtx.createCloneReplicaAndTask());
+    }
+
+    // Before createCloneReplicaAndTask, we need to complete the TabletSchedCtx.
+    // 1. If you generate {tabletId, srcReplica, destBe} in selectAlternativeTablets(), it may be invalid at
+    // this point(it may have a long interval between selectAlternativeTablets & createBalanceTask).
+    // You should check the moves' validation.
+    // 2. If you want to generate {srcReplica, destBe} here, just do it.
+    // 3. You should check the path slots of src & dest.
+    protected abstract void completeSchedCtx(TabletSchedCtx tabletCtx, Map<Long, PathSlot> backendsWorkingSlots)
+            throws SchedException;
+
+    public Long getToDeleteReplicaId(Long tabletId) {
+        return -1L;
+    }
+
+    public void updateLoadStatistic(Map<String, ClusterLoadStatistic> statisticMap) {
+        this.statisticMap = statisticMap;
+    }
+}
diff --git a/fe/fe-core/src/main/java/org/apache/doris/clone/TabletScheduler.java b/fe/fe-core/src/main/java/org/apache/doris/clone/TabletScheduler.java
index 3720587..9a2cfd6 100644
--- a/fe/fe-core/src/main/java/org/apache/doris/clone/TabletScheduler.java
+++ b/fe/fe-core/src/main/java/org/apache/doris/clone/TabletScheduler.java
@@ -72,15 +72,15 @@ import java.util.stream.Collectors;
 /**
  * TabletScheduler saved the tablets produced by TabletChecker and try to schedule them.
  * It also try to balance the cluster load.
- * 
+ *
  * We are expecting an efficient way to recovery the entire cluster and make it balanced.
  * Case 1:
  *  A Backend is down. All tablets which has replica on this BE should be repaired as soon as possible.
- *  
+ *
  * Case 1.1:
  *  As Backend is down, some tables should be repaired in high priority. So the clone task should be able
  *  to preempted.
- *  
+ *
  * Case 2:
  *  A new Backend is added to the cluster. Replicas should be transfer to that host to balance the cluster load.
  */
@@ -103,10 +103,10 @@ public class TabletScheduler extends MasterDaemon {
      * handling a tablet.
      * Tablet' id can only be removed after the clone task is done(timeout, cancelled or finished).
      * So if a tablet's id is still in allTabletIds, TabletChecker can not add tablet to TabletScheduler.
-     * 
+     *
      * pendingTablets + runningTablets = allTabletIds
-     * 
-     * pendingTablets, allTabletIds, runningTablets and schedHistory are protected by 'synchronized' 
+     *
+     * pendingTablets, allTabletIds, runningTablets and schedHistory are protected by 'synchronized'
      */
     private PriorityQueue<TabletSchedCtx> pendingTablets = new PriorityQueue<>();
     private Set<Long> allTabletIds = Sets.newHashSet();
@@ -120,7 +120,7 @@ public class TabletScheduler extends MasterDaemon {
     // cluster name -> load statistic
     private Map<String, ClusterLoadStatistic> statisticMap = Maps.newConcurrentMap();
     private long lastStatUpdateTime = 0;
-    
+
     private long lastSlotAdjustTime = 0;
 
     private Catalog catalog;
@@ -128,7 +128,8 @@ public class TabletScheduler extends MasterDaemon {
     private TabletInvertedIndex invertedIndex;
     private ColocateTableIndex colocateTableIndex;
     private TabletSchedulerStat stat;
-    
+    private Rebalancer rebalancer;
+
     // result of adding a tablet to pendingTablets
     public enum AddResult {
         ADDED, // success to add
@@ -144,6 +145,7 @@ public class TabletScheduler extends MasterDaemon {
         this.invertedIndex = invertedIndex;
         this.colocateTableIndex = catalog.getColocateTableIndex();
         this.stat = stat;
+        this.rebalancer = new BeLoadRebalancer(infoService, invertedIndex);
     }
 
     public TabletSchedulerStat getStat() {
@@ -209,7 +211,7 @@ public class TabletScheduler extends MasterDaemon {
         if (!force && containsTablet(tablet.getTabletId())) {
             return AddResult.ALREADY_IN;
         }
-        
+
         // if this is not a BALANCE task, and not a force add,
         // and number of scheduling tablets exceed the limit,
         // refuse to add.
@@ -249,7 +251,7 @@ public class TabletScheduler extends MasterDaemon {
      * Then, it will schedule the tablets in pendingTablets.
      * Thirdly, it will check the current running tasks.
      * Finally, it try to balance the cluster if possible.
-     * 
+     *
      * Schedule rules:
      * 1. tablet with higher priority will be scheduled first.
      * 2. high priority should be downgraded if it fails to be schedule too many times.
@@ -283,6 +285,8 @@ public class TabletScheduler extends MasterDaemon {
         }
 
         updateClusterLoadStatistic();
+        rebalancer.updateLoadStatistic(statisticMap);
+
         adjustPriorities();
 
         lastStatUpdateTime = System.currentTimeMillis();
@@ -340,7 +344,7 @@ public class TabletScheduler extends MasterDaemon {
      * 1. in runningTablets with state RUNNING, if being scheduled success.
      * 2. or in schedHistory with state CANCELLING, if some unrecoverable error happens.
      * 3. or in pendingTablets with state PENDING, if failed to be scheduled.
-     * 
+     *
      * if in schedHistory, it should be removed from allTabletIds.
      */
     private void schedulePendingTablets() {
@@ -509,7 +513,7 @@ public class TabletScheduler extends MasterDaemon {
                 throw new SchedException(Status.UNRECOVERABLE, "table's state is not NORMAL");
             }
 
-            if (statusPair.first != TabletStatus.VERSION_INCOMPLETE  
+            if (statusPair.first != TabletStatus.VERSION_INCOMPLETE
                     && (partition.getState() != PartitionState.NORMAL || tableState != OlapTableState.NORMAL)
                     && tableState != OlapTableState.WAITING_STABLE) {
                 // If table is under ALTER process(before FINISHING), do not allow to add or delete replica.
@@ -599,7 +603,7 @@ public class TabletScheduler extends MasterDaemon {
      * 2. find an appropriate source replica:
      *      1. source replica should be healthy
      *      2. backend of source replica has available slot for clone.
-     *      
+     *
      * 3. send clone task to destination backend
      */
     private void handleReplicaMissing(TabletSchedCtx tabletCtx, AgentBatchTask batchTask) throws SchedException {
@@ -619,7 +623,7 @@ public class TabletScheduler extends MasterDaemon {
     /**
      * Replica version is incomplete, which means this replica is missing some version,
      * and need to be cloned from a healthy replica, in-place.
-     * 
+     *
      * 1. find the incomplete replica as destination replica
      * 2. find a healthy replica as source replica
      * 3. send clone task
@@ -662,7 +666,7 @@ public class TabletScheduler extends MasterDaemon {
      * under decommission.
      * First, we try to find a version incomplete replica on available BE.
      * If failed to find, then try to find a new BE to clone the replicas.
-     * 
+     *
      * Give examples of why:
      * Tablet X has 3 replicas on A, B, C 3 BEs.
      * C is decommission, so we choose the BE D to relocating the new replica,
@@ -708,7 +712,8 @@ public class TabletScheduler extends MasterDaemon {
      *  5. replica's last failed version > 0
      *  6. replica with lower version
      *  7. replica not in right cluster
-     *  8. replica in higher load backend
+     *  8. replica is the src replica of a rebalance task, we can try to get it from rebalancer
+     *  9. replica on higher load backend
      */
     private void handleRedundantReplica(TabletSchedCtx tabletCtx, boolean force) throws SchedException {
         stat.counterReplicaRedundantErr.incrementAndGet();
@@ -721,6 +726,7 @@ public class TabletScheduler extends MasterDaemon {
                 || deleteReplicaWithLowerVersion(tabletCtx, force)
                 || deleteReplicaOnSameHost(tabletCtx, force)
                 || deleteReplicaNotInCluster(tabletCtx, force)
+                || deleteReplicaChosenByRebalancer(tabletCtx, force)
                 || deleteReplicaOnHighLoadBackend(tabletCtx, force)) {
             // if we delete at least one redundant replica, we still throw a SchedException with status FINISHED
             // to remove this tablet from the pendingTablets(consider it as finished)
@@ -845,12 +851,25 @@ public class TabletScheduler extends MasterDaemon {
         return false;
     }
 
+    private boolean deleteReplicaChosenByRebalancer(TabletSchedCtx tabletCtx, boolean force) throws SchedException {
+        Long id = rebalancer.getToDeleteReplicaId(tabletCtx.getTabletId());
+        if (id == -1L) {
+            return false;
+        }
+        Replica chosenReplica = tabletCtx.getTablet().getReplicaById(id);
+        if (chosenReplica != null) {
+            deleteReplicaInternal(tabletCtx, chosenReplica, "src replica of rebalance", force);
+            return true;
+        }
+        return false;
+    }
+
     private boolean deleteReplicaOnHighLoadBackend(TabletSchedCtx tabletCtx, boolean force) throws SchedException {
         ClusterLoadStatistic statistic = statisticMap.get(tabletCtx.getCluster());
         if (statistic == null) {
             return false;
         }
-        
+
         return deleteFromHighLoadBackend(tabletCtx, tabletCtx.getReplicas(), force, statistic);
     }
 
@@ -869,7 +888,7 @@ public class TabletScheduler extends MasterDaemon {
              * sure that at least one replica can be chosen.
              * This can happen if the Doris cluster is deployed with all, for example, SSD medium,
              * but create all tables with HDD storage medium property. Then getLoadScore(SSD) will
-             * always return 0.0, so that no replica will be chosen. 
+             * always return 0.0, so that no replica will be chosen.
              */
             double loadScore = 0.0;
             if (beStatistic.hasMedium(tabletCtx.getStorageMedium())) {
@@ -937,7 +956,7 @@ public class TabletScheduler extends MasterDaemon {
                 throw new SchedException(Status.UNRECOVERABLE, e.getMessage());
             }
         }
-        
+
         // delete this replica from catalog.
         // it will also delete replica from tablet inverted index.
         tabletCtx.deleteReplica(replica);
@@ -977,7 +996,7 @@ public class TabletScheduler extends MasterDaemon {
      * Cluster migration, which means the tablet has enough healthy replicas,
      * but some replicas are not in right cluster.
      * It is just same as 'replica missing'.
-     * 
+     *
      * after clone finished, the replica in wrong cluster will be treated as redundant, and will be deleted soon.
      */
     private void handleReplicaClusterMigration(TabletSchedCtx tabletCtx, AgentBatchTask batchTask)
@@ -990,10 +1009,10 @@ public class TabletScheduler extends MasterDaemon {
      * Replicas of colocate table's tablet does not locate on right backends set.
      *      backends set:       1,2,3
      *      tablet replicas:    1,2,5
-     *      
+     *
      *      backends set:       1,2,3
      *      tablet replicas:    1,2
-     *      
+     *
      *      backends set:       1,2,3
      *      tablet replicas:    1,2,4,5
      */
@@ -1022,7 +1041,7 @@ public class TabletScheduler extends MasterDaemon {
             LOG.info("balance is disabled. skip selecting tablets for balance");
             return;
         }
-        
+
         long numOfBalancingTablets = getBalanceTabletsNumber();
         if (numOfBalancingTablets > Config.max_balancing_tablets) {
             LOG.info("number of balancing tablets {} exceed limit: {}, skip selecting tablets for balance",
@@ -1030,8 +1049,7 @@ public class TabletScheduler extends MasterDaemon {
             return;
         }
 
-        LoadBalancer loadBalancer = new LoadBalancer(statisticMap);
-        List<TabletSchedCtx> alternativeTablets = loadBalancer.selectAlternativeTablets();
+        List<TabletSchedCtx> alternativeTablets = rebalancer.selectAlternativeTablets();
         for (TabletSchedCtx tabletCtx : alternativeTablets) {
             addTablet(tabletCtx, false);
         }
@@ -1042,8 +1060,7 @@ public class TabletScheduler extends MasterDaemon {
      */
     private void doBalance(TabletSchedCtx tabletCtx, AgentBatchTask batchTask) throws SchedException {
         stat.counterBalanceSchedule.incrementAndGet();
-        LoadBalancer loadBalancer = new LoadBalancer(statisticMap);
-        loadBalancer.createBalanceTask(tabletCtx, backendsWorkingSlots, batchTask);
+        rebalancer.createBalanceTask(tabletCtx, backendsWorkingSlots, batchTask);
     }
 
     // choose a path on a backend which is fit for the tablet
@@ -1128,7 +1145,7 @@ public class TabletScheduler extends MasterDaemon {
                 return rootPathLoadStatistic;
             }
         }
-        
+
         throw new SchedException(Status.SCHEDULE_FAILED, "unable to find dest path which can be fit in");
     }
 
@@ -1237,7 +1254,7 @@ public class TabletScheduler extends MasterDaemon {
 
     /**
      * Gather the running statistic of the task.
-     * It will be evaluated for future strategy.  
+     * It will be evaluated for future strategy.
      * This should only be called when the tablet is down with state FINISHED.
      */
     private void gatherStatistics(TabletSchedCtx tabletCtx) {
@@ -1274,7 +1291,7 @@ public class TabletScheduler extends MasterDaemon {
      * We should finished the task if
      * 1. Tablet is already healthy
      * 2. Task is timeout.
-     * 
+     *
      * But here we just handle the timeout case here. Let the 'finishCloneTask()' check if tablet is healthy.
      * We guarantee that if tablet is in runningTablets, the 'finishCloneTask()' will finally be called,
      * so no need to worry that running tablets will never end.
@@ -1292,7 +1309,7 @@ public class TabletScheduler extends MasterDaemon {
                 removeTabletCtx(tabletSchedCtx, "timeout");
             }
         }
-        
+
         // 2. release ctx
         timeoutTablets.stream().forEach(t -> {
             releaseTabletCtx(t, TabletSchedCtx.State.CANCELLED);


---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@doris.apache.org
For additional commands, e-mail: commits-help@doris.apache.org