You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@mesos.apache.org by mz...@apache.org on 2019/04/25 21:03:20 UTC

[mesos] 03/04: Fixed a bug in the random sorter.

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

mzhu pushed a commit to branch 1.8.x
in repository https://gitbox.apache.org/repos/asf/mesos.git

commit e6ca1a6942e999a331cf0a3fe33d0f045272050a
Author: Meng Zhu <mz...@mesosphere.io>
AuthorDate: Wed Apr 24 10:51:06 2019 -0700

    Fixed a bug in the random sorter.
    
    Currently, in the presence of hierarchical roles, the
    random sorter shuffles roles level by level and then pick
    the active leave nodes using DFS. This could generate
    non-uniform random result since active leaves in a subtree
    are always picked together.
    
    This patch fixes the issue by first calculating the relative
    weights of each active leaf node and shuffle all of them
    only once.
    
    Review: https://reviews.apache.org/r/70429
---
 src/master/allocator/sorter/random/sorter.cpp | 131 ++++++++++++++------------
 src/master/allocator/sorter/random/sorter.hpp |  29 ++++++
 2 files changed, 102 insertions(+), 58 deletions(-)

diff --git a/src/master/allocator/sorter/random/sorter.cpp b/src/master/allocator/sorter/random/sorter.cpp
index ad06b0b..8cf01e3 100644
--- a/src/master/allocator/sorter/random/sorter.cpp
+++ b/src/master/allocator/sorter/random/sorter.cpp
@@ -33,6 +33,7 @@
 #include <stout/option.hpp>
 #include <stout/strings.hpp>
 
+using std::pair;
 using std::set;
 using std::string;
 using std::vector;
@@ -460,66 +461,16 @@ void RandomSorter::remove(const SlaveID& slaveId, const Resources& resources)
 
 vector<string> RandomSorter::sort()
 {
-  std::function<void (Node*)> shuffleTree = [this, &shuffleTree](Node* node) {
-    // Inactive leaves are always stored at the end of the
-    // `children` vector; this means that we should only shuffle
-    // the prefix of the vector before the first inactive leaf.
-    auto inactiveBegin = std::find_if(
-        node->children.begin(),
-        node->children.end(),
-        [](Node* n) { return n->kind == Node::INACTIVE_LEAF; });
-
-    vector<double> weights(inactiveBegin - node->children.begin());
-
-    for (int i = 0; i < inactiveBegin - node->children.begin(); ++i) {
-      weights[i] = getWeight(node->children[i]);
-    }
-
-    weightedShuffle(node->children.begin(), inactiveBegin, weights, generator);
-
-    foreach (Node* child, node->children) {
-      if (child->kind == Node::INTERNAL) {
-        shuffleTree(child);
-      } else if (child->kind == Node::INACTIVE_LEAF) {
-        break;
-      }
-    }
-  };
-
-  shuffleTree(root);
-
-  // Return all active leaves in the tree via pre-order traversal.
-  // The children of each node are already shuffled, with
-  // inactive leaves stored after active leaves and internal nodes.
-  vector<string> result;
-
-  // TODO(bmahler): This over-reserves where there are inactive
-  // clients, only reserve the number of active clients.
-  result.reserve(clients.size());
-
-  std::function<void (const Node*)> listClients =
-      [&listClients, &result](const Node* node) {
-    foreach (const Node* child, node->children) {
-      switch (child->kind) {
-        case Node::ACTIVE_LEAF:
-          result.push_back(child->clientPath());
-          break;
-
-        case Node::INACTIVE_LEAF:
-          // As soon as we see the first inactive leaf, we can stop
-          // iterating over the current node's list of children.
-          return;
-
-        case Node::INTERNAL:
-          listClients(child);
-          break;
-      }
-    }
-  };
+  pair<vector<string>, vector<double>> clientsAndWeights =
+    SortInfo(this).getClientsAndWeights();
 
-  listClients(root);
+  weightedShuffle(
+      clientsAndWeights.first.begin(),
+      clientsAndWeights.first.end(),
+      clientsAndWeights.second,
+      generator);
 
-  return result;
+  return std::move(clientsAndWeights.first);
 }
 
 
@@ -599,6 +550,70 @@ RandomSorter::Node* RandomSorter::find(const string& clientPath) const
   return client;
 }
 
+
+pair<vector<string>, vector<double>>
+RandomSorter::SortInfo::getClientsAndWeights()
+{
+  updateRelativeWeights();
+
+  return std::make_pair(clients, weights);
+}
+
+
+void RandomSorter::SortInfo::updateRelativeWeights()
+{
+  hashset<Node*> activeInternalNodes = sorter->activeInternalNodes();
+
+  auto isActive = [&activeInternalNodes](Node* node) {
+    return node->kind == Node::ACTIVE_LEAF ||
+           activeInternalNodes.contains(node);
+  };
+
+  clients.reserve(sorter->clients.size());
+  weights.reserve(sorter->clients.size());
+
+  // We use the following formula to compute the relative weights (Rw):
+  //
+  //                                    weight(node)
+  // Rw(node) = Rw(parent) * -------------------------------------------
+  //                         weight(node) + SUM(weight(active siblings))
+  //
+  // Using pre-order traversal to calculate node's relative weight.
+  // Active leaves and their relative weights are stored in `clients`
+  // and `weights`.
+  std::function<void(Node*, double, double)> calculateRelativeWeights =
+    [&](Node* node, double siblingWeights, double parentRelativeWeight) {
+      // We only calculate relative weights for active nodes.
+      if (!isActive(node)) {
+        return;
+      }
+      double relativeWeight = parentRelativeWeight * sorter->getWeight(node) /
+                              (sorter->getWeight(node) + siblingWeights);
+
+      // Store the result for active leaves.
+      if (node->kind == Node::ACTIVE_LEAF) {
+        clients.push_back(node->clientPath());
+        weights.push_back(relativeWeight);
+      }
+
+      // Calculate active children's total weights.
+      double totalWeights_ = 0.0;
+
+      foreach (Node* child, node->children) {
+        totalWeights_ += isActive(child) ? sorter->getWeight(child) : 0.0;
+      }
+
+      foreach (Node* child, node->children) {
+        if (isActive(child)) {
+          calculateRelativeWeights(
+              child, totalWeights_ - sorter->getWeight(child), relativeWeight);
+        }
+      }
+    };
+
+  calculateRelativeWeights(sorter->root, 0.0, 1.0);
+}
+
 } // namespace allocator {
 } // namespace master {
 } // namespace internal {
diff --git a/src/master/allocator/sorter/random/sorter.hpp b/src/master/allocator/sorter/random/sorter.hpp
index 0c2d645..15bc75c 100644
--- a/src/master/allocator/sorter/random/sorter.hpp
+++ b/src/master/allocator/sorter/random/sorter.hpp
@@ -133,6 +133,35 @@ private:
   // internal node in the tree (not a client).
   Node* find(const std::string& clientPath) const;
 
+  // Sorting related info are kept in memory to avoid recalculations.
+  //
+  // TODO(mzhu): Keep this in the memory to avoid recalculations.
+  struct SortInfo
+  {
+  public:
+    SortInfo(const RandomSorter* sorter_) : sorter(sorter_) {}
+
+    // Returns a pair of vectors which are active clients and
+    // their corresponding relative weights.
+    std::pair<std::vector<std::string>, std::vector<double>>
+    getClientsAndWeights();
+
+  private:
+    void updateRelativeWeights();
+
+    std::vector<std::string> clients;
+
+    // Relative weights of the `clients` above.
+    // Relative weight denotes the weight of an active leaf node relative to
+    // other active leaf nodes given their configured weights. The number here
+    // stands for the probability of a given node being shuffled to the 1st in
+    // all the nodes in a random shuffle. Intuitively, the sum of all relative
+    // weights should be one.
+    std::vector<double> weights;
+
+    const RandomSorter* sorter;
+  };
+
   // Used for random number generation.
   std::mt19937 generator;