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.master.balancer;
019
020/** An implementation of the {@link org.apache.hadoop.hbase.master.LoadBalancer} that assigns regions
021 * based on the amount they are cached on a given server. A region can move across the region
022 * servers whenever a region server shuts down or crashes. The region server preserves the cache
023 * periodically and restores the cache when it is restarted. This balancer implements a mechanism
024 * where it maintains the amount by which a region is cached on a region server. During balancer
025 * run, a region plan is generated that takes into account this cache information and tries to
026 * move the regions so that the cache minimally impacted.
027 */
028
029import static org.apache.hadoop.hbase.HConstants.BUCKET_CACHE_PERSISTENT_PATH_KEY;
030
031import java.text.DecimalFormat;
032import java.util.ArrayDeque;
033import java.util.ArrayList;
034import java.util.Arrays;
035import java.util.Deque;
036import java.util.HashMap;
037import java.util.List;
038import java.util.Map;
039import java.util.Optional;
040import org.apache.hadoop.conf.Configuration;
041import org.apache.hadoop.hbase.ClusterMetrics;
042import org.apache.hadoop.hbase.RegionMetrics;
043import org.apache.hadoop.hbase.ServerMetrics;
044import org.apache.hadoop.hbase.ServerName;
045import org.apache.hadoop.hbase.Size;
046import org.apache.hadoop.hbase.TableName;
047import org.apache.hadoop.hbase.client.RegionInfo;
048import org.apache.hadoop.hbase.master.RegionPlan;
049import org.apache.hadoop.hbase.util.Pair;
050import org.apache.yetus.audience.InterfaceAudience;
051import org.slf4j.Logger;
052import org.slf4j.LoggerFactory;
053
054@InterfaceAudience.Private
055public class CacheAwareLoadBalancer extends StochasticLoadBalancer {
056  private static final Logger LOG = LoggerFactory.getLogger(CacheAwareLoadBalancer.class);
057
058  public static final String CACHE_RATIO_THRESHOLD =
059    "hbase.master.balancer.stochastic.throttling.cacheRatio";
060  public static final float CACHE_RATIO_THRESHOLD_DEFAULT = 0.8f;
061
062  public Float ratioThreshold;
063
064  private Long sleepTime;
065  private Configuration configuration;
066
067  public enum GeneratorFunctionType {
068    LOAD,
069    CACHE_RATIO
070  }
071
072  @Override
073  public synchronized void loadConf(Configuration configuration) {
074    this.configuration = configuration;
075    this.costFunctions = new ArrayList<>();
076    super.loadConf(configuration);
077    ratioThreshold =
078      this.configuration.getFloat(CACHE_RATIO_THRESHOLD, CACHE_RATIO_THRESHOLD_DEFAULT);
079    sleepTime = configuration.getLong(MOVE_THROTTLING, MOVE_THROTTLING_DEFAULT.toMillis());
080  }
081
082  @Override
083  protected Map<Class<? extends CandidateGenerator>, CandidateGenerator>
084    createCandidateGenerators(Configuration conf) {
085    Map<Class<? extends CandidateGenerator>, CandidateGenerator> candidateGenerators =
086      new HashMap<>(2);
087    candidateGenerators.put(CacheAwareSkewnessCandidateGenerator.class,
088      new CacheAwareSkewnessCandidateGenerator());
089    candidateGenerators.put(CacheAwareCandidateGenerator.class, new CacheAwareCandidateGenerator());
090    return candidateGenerators;
091  }
092
093  @Override
094  protected List<CostFunction> createCostFunctions(Configuration configuration) {
095    List<CostFunction> costFunctions = new ArrayList<>();
096    addCostFunction(costFunctions, new CacheAwareRegionSkewnessCostFunction(configuration));
097    addCostFunction(costFunctions, new CacheAwareCostFunction(configuration));
098    return costFunctions;
099  }
100
101  private void addCostFunction(List<CostFunction> costFunctions, CostFunction costFunction) {
102    if (costFunction.getMultiplier() > 0) {
103      costFunctions.add(costFunction);
104    }
105  }
106
107  @Override
108  public void updateClusterMetrics(ClusterMetrics clusterMetrics) {
109    this.clusterStatus = clusterMetrics;
110    updateRegionLoad();
111  }
112
113  /**
114   * Collect the amount of region cached for all the regions from all the active region servers.
115   */
116  private void updateRegionLoad() {
117    loads = new HashMap<>();
118    regionCacheRatioOnOldServerMap = new HashMap<>();
119    Map<String, Pair<ServerName, Integer>> regionCacheRatioOnCurrentServerMap = new HashMap<>();
120
121    // Build current region cache statistics
122    clusterStatus.getLiveServerMetrics().forEach((ServerName sn, ServerMetrics sm) -> {
123      // Create a map of region and the server where it is currently hosted
124      sm.getRegionMetrics().forEach((byte[] regionName, RegionMetrics rm) -> {
125        String regionEncodedName = RegionInfo.encodeRegionName(regionName);
126
127        Deque<BalancerRegionLoad> rload = new ArrayDeque<>();
128
129        // Get the total size of the hFiles in this region
130        int regionSizeMB = (int) rm.getRegionSizeMB().get(Size.Unit.MEGABYTE);
131
132        rload.add(new BalancerRegionLoad(rm));
133        // Maintain a map of region and it's total size. This is needed to calculate the cache
134        // ratios for the regions cached on old region servers
135        regionCacheRatioOnCurrentServerMap.put(regionEncodedName, new Pair<>(sn, regionSizeMB));
136        loads.put(regionEncodedName, rload);
137      });
138    });
139
140    // Build cache statistics for the regions hosted previously on old region servers
141    clusterStatus.getLiveServerMetrics().forEach((ServerName sn, ServerMetrics sm) -> {
142      // Find if a region was previously hosted on a server other than the one it is currently
143      // hosted on.
144      sm.getRegionCachedInfo().forEach((String regionEncodedName, Integer regionSizeInCache) -> {
145        // If the region is found in regionCacheRatioOnCurrentServerMap, it is currently hosted on
146        // this server
147        if (regionCacheRatioOnCurrentServerMap.containsKey(regionEncodedName)) {
148          ServerName currentServer =
149            regionCacheRatioOnCurrentServerMap.get(regionEncodedName).getFirst();
150          if (!ServerName.isSameAddress(currentServer, sn)) {
151            int regionSizeMB =
152              regionCacheRatioOnCurrentServerMap.get(regionEncodedName).getSecond();
153            float regionCacheRatioOnOldServer =
154              regionSizeMB == 0 ? 0.0f : (float) regionSizeInCache / regionSizeMB;
155            regionCacheRatioOnOldServerMap.put(regionEncodedName,
156              new Pair<>(sn, regionCacheRatioOnOldServer));
157          }
158        }
159      });
160    });
161  }
162
163  private RegionInfo getRegionInfoByEncodedName(BalancerClusterState cluster, String regionName) {
164    Optional<RegionInfo> regionInfoOptional =
165      Arrays.stream(cluster.regions).filter((RegionInfo ri) -> {
166        return regionName.equals(ri.getEncodedName());
167      }).findFirst();
168
169    if (regionInfoOptional.isPresent()) {
170      return regionInfoOptional.get();
171    }
172    return null;
173  }
174
175  @Override
176  public void throttle(RegionPlan plan) {
177    Pair<ServerName, Float> rsRatio = this.regionCacheRatioOnOldServerMap.get(plan.getRegionName());
178    if (
179      rsRatio != null && plan.getDestination().equals(rsRatio.getFirst())
180        && rsRatio.getSecond() >= ratioThreshold
181    ) {
182      LOG.debug("Moving region {} to server {} with cache ratio {}. No throttling needed.",
183        plan.getRegionInfo().getEncodedName(), plan.getDestination(), rsRatio.getSecond());
184    } else {
185      if (rsRatio != null) {
186        LOG.debug("Moving region {} to server {} with cache ratio: {}. Throttling move for {}ms.",
187          plan.getRegionInfo().getEncodedName(), plan.getDestination(),
188          plan.getDestination().equals(rsRatio.getFirst()) ? rsRatio.getSecond() : "unknown",
189          sleepTime);
190      } else {
191        LOG.debug(
192          "Moving region {} to server {} with no cache ratio info for the region. "
193            + "Throttling move for {}ms.",
194          plan.getRegionInfo().getEncodedName(), plan.getDestination(), sleepTime);
195      }
196      try {
197        Thread.sleep(sleepTime);
198      } catch (InterruptedException e) {
199        throw new RuntimeException(e);
200      }
201    }
202  }
203
204  @Override
205  protected List<RegionPlan> balanceTable(TableName tableName,
206    Map<ServerName, List<RegionInfo>> loadOfOneTable) {
207    final Map<String, Pair<ServerName, Float>> snapshot = new HashMap<>();
208    snapshot.putAll(this.regionCacheRatioOnOldServerMap);
209    List<RegionPlan> plans = super.balanceTable(tableName, loadOfOneTable);
210    plans.sort((p1, p2) -> {
211      Pair<ServerName, Float> pair1 = snapshot.get(p1.getRegionName());
212      Float ratio1 =
213        pair1 == null ? 0 : pair1.getFirst().equals(p1.getDestination()) ? pair1.getSecond() : 0f;
214      Pair<ServerName, Float> pair2 = snapshot.get(p2.getRegionName());
215      Float ratio2 =
216        pair2 == null ? 0 : pair2.getFirst().equals(p2.getDestination()) ? pair2.getSecond() : 0f;
217      return ratio1.compareTo(ratio2) * (-1);
218    });
219    return plans;
220  }
221
222  private class CacheAwareCandidateGenerator extends CandidateGenerator {
223    @Override
224    protected BalanceAction generate(BalancerClusterState cluster) {
225      // Move the regions to the servers they were previously hosted on based on the cache ratio
226      if (
227        !regionCacheRatioOnOldServerMap.isEmpty()
228          && regionCacheRatioOnOldServerMap.entrySet().iterator().hasNext()
229      ) {
230        Map.Entry<String, Pair<ServerName, Float>> regionCacheRatioServerMap =
231          regionCacheRatioOnOldServerMap.entrySet().iterator().next();
232        // Get the server where this region was previously hosted
233        String regionEncodedName = regionCacheRatioServerMap.getKey();
234        RegionInfo regionInfo = getRegionInfoByEncodedName(cluster, regionEncodedName);
235        if (regionInfo == null) {
236          LOG.warn("Region {} not found", regionEncodedName);
237          regionCacheRatioOnOldServerMap.remove(regionEncodedName);
238          return BalanceAction.NULL_ACTION;
239        }
240        if (regionInfo.isMetaRegion() || regionInfo.getTable().isSystemTable()) {
241          regionCacheRatioOnOldServerMap.remove(regionEncodedName);
242          return BalanceAction.NULL_ACTION;
243        }
244        int regionIndex = cluster.regionsToIndex.get(regionInfo);
245        int oldServerIndex = cluster.serversToIndex
246          .get(regionCacheRatioOnOldServerMap.get(regionEncodedName).getFirst().getAddress());
247        if (oldServerIndex < 0) {
248          LOG.warn("Server previously hosting region {} not found", regionEncodedName);
249          regionCacheRatioOnOldServerMap.remove(regionEncodedName);
250          return BalanceAction.NULL_ACTION;
251        }
252
253        float oldRegionCacheRatio =
254          cluster.getOrComputeRegionCacheRatio(regionIndex, oldServerIndex);
255        int currentServerIndex = cluster.regionIndexToServerIndex[regionIndex];
256        float currentRegionCacheRatio =
257          cluster.getOrComputeRegionCacheRatio(regionIndex, currentServerIndex);
258
259        BalanceAction action = generatePlan(cluster, regionIndex, currentServerIndex,
260          currentRegionCacheRatio, oldServerIndex, oldRegionCacheRatio);
261        regionCacheRatioOnOldServerMap.remove(regionEncodedName);
262        return action;
263      }
264      return BalanceAction.NULL_ACTION;
265    }
266
267    private BalanceAction generatePlan(BalancerClusterState cluster, int regionIndex,
268      int currentServerIndex, float cacheRatioOnCurrentServer, int oldServerIndex,
269      float cacheRatioOnOldServer) {
270      return moveRegionToOldServer(cluster, regionIndex, currentServerIndex,
271        cacheRatioOnCurrentServer, oldServerIndex, cacheRatioOnOldServer)
272          ? getAction(currentServerIndex, regionIndex, oldServerIndex, -1)
273          : BalanceAction.NULL_ACTION;
274    }
275
276    private boolean moveRegionToOldServer(BalancerClusterState cluster, int regionIndex,
277      int currentServerIndex, float cacheRatioOnCurrentServer, int oldServerIndex,
278      float cacheRatioOnOldServer) {
279      // Find if the region has already moved by comparing the current server index with the
280      // current server index. This can happen when other candidate generator has moved the region
281      if (currentServerIndex < 0 || oldServerIndex < 0) {
282        return false;
283      }
284
285      DecimalFormat df = new DecimalFormat("#");
286      df.setMaximumFractionDigits(4);
287
288      float cacheRatioDiffThreshold = 0.6f;
289
290      // Conditions for moving the region
291
292      // If the region is fully cached on the old server, move the region back
293      if (cacheRatioOnOldServer == 1.0f) {
294        if (LOG.isDebugEnabled()) {
295          LOG.debug("Region {} moved to the old server {} as it is fully cached there",
296            cluster.regions[regionIndex].getEncodedName(), cluster.servers[oldServerIndex]);
297        }
298        return true;
299      }
300
301      // Move the region back to the old server if it is cached equally on both the servers
302      if (cacheRatioOnCurrentServer == cacheRatioOnOldServer) {
303        if (LOG.isDebugEnabled()) {
304          LOG.debug(
305            "Region {} moved from {} to {} as the region is cached {} equally on both servers",
306            cluster.regions[regionIndex].getEncodedName(), cluster.servers[currentServerIndex],
307            cluster.servers[oldServerIndex], df.format(cacheRatioOnCurrentServer));
308        }
309        return true;
310      }
311
312      // If the region is not fully cached on either of the servers, move the region back to the
313      // old server if the region cache ratio on the current server is still much less than the old
314      // server
315      if (
316        cacheRatioOnOldServer > 0.0f
317          && cacheRatioOnCurrentServer / cacheRatioOnOldServer < cacheRatioDiffThreshold
318      ) {
319        if (LOG.isDebugEnabled()) {
320          LOG.debug(
321            "Region {} moved from {} to {} as region cache ratio {} is better than the current "
322              + "cache ratio {}",
323            cluster.regions[regionIndex].getEncodedName(), cluster.servers[currentServerIndex],
324            cluster.servers[oldServerIndex], cacheRatioOnCurrentServer,
325            df.format(cacheRatioOnCurrentServer));
326        }
327        return true;
328      }
329
330      if (LOG.isDebugEnabled()) {
331        LOG.debug(
332          "Region {} not moved from {} to {} with current cache ratio {} and old cache ratio {}",
333          cluster.regions[regionIndex], cluster.servers[currentServerIndex],
334          cluster.servers[oldServerIndex], cacheRatioOnCurrentServer,
335          df.format(cacheRatioOnCurrentServer));
336      }
337      return false;
338    }
339  }
340
341  private class CacheAwareSkewnessCandidateGenerator extends LoadCandidateGenerator {
342    @Override
343    BalanceAction pickRandomRegions(BalancerClusterState cluster, int thisServer, int otherServer) {
344      // First move all the regions which were hosted previously on some other server back to their
345      // old servers
346      if (
347        !regionCacheRatioOnOldServerMap.isEmpty()
348          && regionCacheRatioOnOldServerMap.entrySet().iterator().hasNext()
349      ) {
350        // Get the first region index in the historical cache ratio list
351        Map.Entry<String, Pair<ServerName, Float>> regionEntry =
352          regionCacheRatioOnOldServerMap.entrySet().iterator().next();
353        String regionEncodedName = regionEntry.getKey();
354
355        RegionInfo regionInfo = getRegionInfoByEncodedName(cluster, regionEncodedName);
356        if (regionInfo == null) {
357          LOG.warn("Region {} does not exist", regionEncodedName);
358          regionCacheRatioOnOldServerMap.remove(regionEncodedName);
359          return BalanceAction.NULL_ACTION;
360        }
361        if (regionInfo.isMetaRegion() || regionInfo.getTable().isSystemTable()) {
362          regionCacheRatioOnOldServerMap.remove(regionEncodedName);
363          return BalanceAction.NULL_ACTION;
364        }
365
366        int regionIndex = cluster.regionsToIndex.get(regionInfo);
367
368        // Get the current host name for this region
369        thisServer = cluster.regionIndexToServerIndex[regionIndex];
370
371        // Get the old server index
372        otherServer = cluster.serversToIndex.get(regionEntry.getValue().getFirst().getAddress());
373
374        regionCacheRatioOnOldServerMap.remove(regionEncodedName);
375
376        if (otherServer < 0) {
377          // The old server has been moved to other host and hence, the region cannot be moved back
378          // to the old server
379          if (LOG.isDebugEnabled()) {
380            LOG.debug(
381              "CacheAwareSkewnessCandidateGenerator: Region {} not moved to the old "
382                + "server {} as the server does not exist",
383              regionEncodedName, regionEntry.getValue().getFirst().getHostname());
384          }
385          return BalanceAction.NULL_ACTION;
386        }
387
388        if (LOG.isDebugEnabled()) {
389          LOG.debug(
390            "CacheAwareSkewnessCandidateGenerator: Region {} moved from {} to {} as it "
391              + "was hosted their earlier",
392            regionEncodedName, cluster.servers[thisServer].getHostname(),
393            cluster.servers[otherServer].getHostname());
394        }
395
396        return getAction(thisServer, regionIndex, otherServer, -1);
397      }
398
399      if (thisServer < 0 || otherServer < 0) {
400        return BalanceAction.NULL_ACTION;
401      }
402
403      int regionIndexToMove = pickLeastCachedRegion(cluster, thisServer);
404      if (regionIndexToMove < 0) {
405        if (LOG.isDebugEnabled()) {
406          LOG.debug("CacheAwareSkewnessCandidateGenerator: No region found for movement");
407        }
408        return BalanceAction.NULL_ACTION;
409      }
410      if (LOG.isDebugEnabled()) {
411        LOG.debug(
412          "CacheAwareSkewnessCandidateGenerator: Region {} moved from {} to {} as it is "
413            + "least cached on current server",
414          cluster.regions[regionIndexToMove].getEncodedName(),
415          cluster.servers[thisServer].getHostname(), cluster.servers[otherServer].getHostname());
416      }
417      return getAction(thisServer, regionIndexToMove, otherServer, -1);
418    }
419
420    private int pickLeastCachedRegion(BalancerClusterState cluster, int thisServer) {
421      float minCacheRatio = Float.MAX_VALUE;
422      int leastCachedRegion = -1;
423      for (int i = 0; i < cluster.regionsPerServer[thisServer].length; i++) {
424        int regionIndex = cluster.regionsPerServer[thisServer][i];
425
426        float cacheRatioOnCurrentServer =
427          cluster.getOrComputeRegionCacheRatio(regionIndex, thisServer);
428        if (cacheRatioOnCurrentServer < minCacheRatio) {
429          minCacheRatio = cacheRatioOnCurrentServer;
430          leastCachedRegion = regionIndex;
431        }
432      }
433      return leastCachedRegion;
434    }
435  }
436
437  static class CacheAwareRegionSkewnessCostFunction extends CostFunction {
438    static final String REGION_COUNT_SKEW_COST_KEY =
439      "hbase.master.balancer.stochastic.regionCountCost";
440    static final float DEFAULT_REGION_COUNT_SKEW_COST = 20;
441    private final DoubleArrayCost cost = new DoubleArrayCost();
442
443    CacheAwareRegionSkewnessCostFunction(Configuration conf) {
444      // Load multiplier should be the greatest as it is the most general way to balance data.
445      this.setMultiplier(conf.getFloat(REGION_COUNT_SKEW_COST_KEY, DEFAULT_REGION_COUNT_SKEW_COST));
446    }
447
448    @Override
449    void prepare(BalancerClusterState cluster) {
450      super.prepare(cluster);
451      cost.prepare(cluster.numServers);
452      cost.applyCostsChange(costs -> {
453        for (int i = 0; i < cluster.numServers; i++) {
454          costs[i] = cluster.regionsPerServer[i].length;
455        }
456      });
457    }
458
459    @Override
460    protected double cost() {
461      return cost.cost();
462    }
463
464    @Override
465    protected void regionMoved(int region, int oldServer, int newServer) {
466      cost.applyCostsChange(costs -> {
467        costs[oldServer] = cluster.regionsPerServer[oldServer].length;
468        costs[newServer] = cluster.regionsPerServer[newServer].length;
469      });
470    }
471
472    @Override
473    public final void updateWeight(Map<Class<? extends CandidateGenerator>, Double> weights) {
474      weights.merge(LoadCandidateGenerator.class, cost(), Double::sum);
475    }
476  }
477
478  static class CacheAwareCostFunction extends CostFunction {
479    private static final String CACHE_COST_KEY = "hbase.master.balancer.stochastic.cacheCost";
480    private double cacheRatio;
481    private double bestCacheRatio;
482
483    private static final float DEFAULT_CACHE_COST = 20;
484
485    CacheAwareCostFunction(Configuration conf) {
486      boolean isPersistentCache = conf.get(BUCKET_CACHE_PERSISTENT_PATH_KEY) != null;
487      // Disable the CacheAwareCostFunction if the cached file list persistence is not enabled
488      this.setMultiplier(
489        !isPersistentCache ? 0.0f : conf.getFloat(CACHE_COST_KEY, DEFAULT_CACHE_COST));
490      bestCacheRatio = 0.0;
491      cacheRatio = 0.0;
492    }
493
494    @Override
495    void prepare(BalancerClusterState cluster) {
496      super.prepare(cluster);
497      cacheRatio = 0.0;
498      bestCacheRatio = 0.0;
499
500      for (int region = 0; region < cluster.numRegions; region++) {
501        cacheRatio += cluster.getOrComputeWeightedRegionCacheRatio(region,
502          cluster.regionIndexToServerIndex[region]);
503        bestCacheRatio += cluster.getOrComputeWeightedRegionCacheRatio(region,
504          getServerWithBestCacheRatioForRegion(region));
505      }
506
507      cacheRatio = bestCacheRatio == 0 ? 1.0 : cacheRatio / bestCacheRatio;
508      if (LOG.isDebugEnabled()) {
509        LOG.debug("CacheAwareCostFunction: Cost: {}", 1 - cacheRatio);
510      }
511    }
512
513    @Override
514    protected double cost() {
515      return scale(0, 1, 1 - cacheRatio);
516    }
517
518    @Override
519    protected void regionMoved(int region, int oldServer, int newServer) {
520      double regionCacheRatioOnOldServer =
521        cluster.getOrComputeWeightedRegionCacheRatio(region, oldServer);
522      double regionCacheRatioOnNewServer =
523        cluster.getOrComputeWeightedRegionCacheRatio(region, newServer);
524      double cacheRatioDiff = regionCacheRatioOnNewServer - regionCacheRatioOnOldServer;
525      double normalizedDelta = bestCacheRatio == 0.0 ? 0.0 : cacheRatioDiff / bestCacheRatio;
526      cacheRatio += normalizedDelta;
527      if (LOG.isDebugEnabled() && (cacheRatio < 0.0 || cacheRatio > 1.0)) {
528        LOG.debug(
529          "CacheAwareCostFunction:regionMoved:region:{}:from:{}:to:{}:regionCacheRatioOnOldServer:{}:"
530            + "regionCacheRatioOnNewServer:{}:bestRegionCacheRatio:{}:cacheRatio:{}",
531          cluster.regions[region].getEncodedName(), cluster.servers[oldServer].getHostname(),
532          cluster.servers[newServer].getHostname(), regionCacheRatioOnOldServer,
533          regionCacheRatioOnNewServer, bestCacheRatio, cacheRatio);
534      }
535    }
536
537    private int getServerWithBestCacheRatioForRegion(int region) {
538      return cluster.getOrComputeServerWithBestRegionCachedRatio()[region];
539    }
540
541    @Override
542    public void updateWeight(Map<Class<? extends CandidateGenerator>, Double> weights) {
543      weights.merge(LoadCandidateGenerator.class, cost(), Double::sum);
544    }
545  }
546}