summaryrefslogtreecommitdiff
diff options
context:
space:
mode:
authorKaloian Manassiev <kaloian.manassiev@mongodb.com>2016-05-05 16:43:53 -0400
committerKaloian Manassiev <kaloian.manassiev@mongodb.com>2016-05-09 10:07:16 -0400
commit4176a6eadb2598c54862d7d80369febf7e1ecce9 (patch)
tree8e7ed2da80984030bf7bf88ac59f2213cd9ee7f0
parentb31ca56158bb6bf70f5037e2d2a9403fb4558c04 (diff)
downloadmongo-4176a6eadb2598c54862d7d80369febf7e1ecce9.tar.gz
SERVER-24071 Introduce ChunkRange class
This change introduces a ChunkRange class to be used for parsing and serialization of chunk min and max bounds. It also switches the chunk manager and sharding utilities to use ranges encoding the chunk bounds separately.
-rw-r--r--jstests/sharding/mongos_validate_writes.js128
-rw-r--r--src/mongo/dbtests/chunktests.cpp2
-rw-r--r--src/mongo/s/catalog/type_chunk.cpp63
-rw-r--r--src/mongo/s/catalog/type_chunk.h42
-rw-r--r--src/mongo/s/chunk.cpp10
-rw-r--r--src/mongo/s/chunk.h7
-rw-r--r--src/mongo/s/chunk_manager.cpp188
-rw-r--r--src/mongo/s/chunk_manager.h90
-rw-r--r--src/mongo/s/shard_util.cpp42
-rw-r--r--src/mongo/s/shard_util.h3
10 files changed, 253 insertions, 322 deletions
diff --git a/jstests/sharding/mongos_validate_writes.js b/jstests/sharding/mongos_validate_writes.js
index 34d40b61172..a217ee04a7f 100644
--- a/jstests/sharding/mongos_validate_writes.js
+++ b/jstests/sharding/mongos_validate_writes.js
@@ -3,89 +3,85 @@
//
// Note that this is *unsafe* with broadcast removes and updates
//
+(function() {
+ 'use strict';
-var st = new ShardingTest({shards: 2, mongos: 3, other: {shardOptions: {verbose: 2}}});
-st.stopBalancer();
+ var st = new ShardingTest({shards: 2, mongos: 3, other: {shardOptions: {verbose: 2}}});
-var mongos = st.s0;
-var staleMongosA = st.s1;
-var staleMongosB = st.s2;
+ var mongos = st.s0;
+ var staleMongosA = st.s1;
+ var staleMongosB = st.s2;
-// Additional logging
-printjson(mongos.getDB("admin").runCommand({setParameter: 1, logLevel: 2}));
-printjson(staleMongosA.getDB("admin").runCommand({setParameter: 1, logLevel: 2}));
-printjson(staleMongosB.getDB("admin").runCommand({setParameter: 1, logLevel: 2}));
-printjson(st._connections[0].getDB("admin").runCommand({setParameter: 1, logLevel: 2}));
-printjson(st._connections[1].getDB("admin").runCommand({setParameter: 1, logLevel: 2}));
+ var admin = mongos.getDB("admin");
+ var config = mongos.getDB("config");
+ var coll = mongos.getCollection("foo.bar");
+ var staleCollA = staleMongosA.getCollection(coll + "");
+ var staleCollB = staleMongosB.getCollection(coll + "");
-var admin = mongos.getDB("admin");
-var config = mongos.getDB("config");
-var coll = mongos.getCollection("foo.bar");
-var staleCollA = staleMongosA.getCollection(coll + "");
-var staleCollB = staleMongosB.getCollection(coll + "");
+ assert.commandWorked(admin.runCommand({enableSharding: coll.getDB() + ""}));
+ st.ensurePrimaryShard(coll.getDB().getName(), 'shard0001');
+ coll.ensureIndex({a: 1});
+ assert.commandWorked(admin.runCommand({shardCollection: coll + "", key: {a: 1}}));
-printjson(admin.runCommand({enableSharding: coll.getDB() + ""}));
-st.ensurePrimaryShard(coll.getDB().getName(), 'shard0001');
-coll.ensureIndex({a: 1});
-printjson(admin.runCommand({shardCollection: coll + "", key: {a: 1}}));
+ // Let the stale mongos see the collection state
+ staleCollA.findOne();
+ staleCollB.findOne();
-// Let the stale mongos see the collection state
-staleCollA.findOne();
-staleCollB.findOne();
+ // Change the collection sharding state
+ coll.drop();
+ coll.ensureIndex({b: 1});
+ assert.commandWorked(admin.runCommand({shardCollection: coll + "", key: {b: 1}}));
-// Change the collection sharding state
-coll.drop();
-coll.ensureIndex({b: 1});
-printjson(admin.runCommand({shardCollection: coll + "", key: {b: 1}}));
+ // Make sure that we can successfully insert, even though we have stale state
+ assert.writeOK(staleCollA.insert({b: "b"}));
-// Make sure that we can successfully insert, even though we have stale state
-assert.writeOK(staleCollA.insert({b: "b"}));
+ // Make sure we unsuccessfully insert with old info
+ assert.writeError(staleCollB.insert({a: "a"}));
-// Make sure we unsuccessfully insert with old info
-assert.writeError(staleCollB.insert({a: "a"}));
+ // Change the collection sharding state
+ coll.drop();
+ coll.ensureIndex({c: 1});
+ assert.commandWorked(admin.runCommand({shardCollection: coll + "", key: {c: 1}}));
-// Change the collection sharding state
-coll.drop();
-coll.ensureIndex({c: 1});
-printjson(admin.runCommand({shardCollection: coll + "", key: {c: 1}}));
+ // Make sure we can successfully upsert, even though we have stale state
+ assert.writeOK(staleCollA.update({c: "c"}, {c: "c"}, true));
-// Make sure we can successfully upsert, even though we have stale state
-assert.writeOK(staleCollA.update({c: "c"}, {c: "c"}, true));
+ // Make sure we unsuccessfully upsert with old info
+ assert.writeError(staleCollB.update({b: "b"}, {b: "b"}, true));
-// Make sure we unsuccessfully upsert with old info
-assert.writeError(staleCollB.update({b: "b"}, {b: "b"}, true));
+ // Change the collection sharding state
+ coll.drop();
+ coll.ensureIndex({d: 1});
+ assert.commandWorked(admin.runCommand({shardCollection: coll + "", key: {d: 1}}));
-// Change the collection sharding state
-coll.drop();
-coll.ensureIndex({d: 1});
-printjson(admin.runCommand({shardCollection: coll + "", key: {d: 1}}));
+ // Make sure we can successfully update, even though we have stale state
+ assert.writeOK(coll.insert({d: "d"}));
-// Make sure we can successfully update, even though we have stale state
-assert.writeOK(coll.insert({d: "d"}));
+ assert.writeOK(staleCollA.update({d: "d"}, {$set: {x: "x"}}, false, false));
+ assert.eq(staleCollA.findOne().x, "x");
-assert.writeOK(staleCollA.update({d: "d"}, {$set: {x: "x"}}, false, false));
-assert.eq(staleCollA.findOne().x, "x");
+ // Make sure we unsuccessfully update with old info
+ assert.writeError(staleCollB.update({c: "c"}, {$set: {x: "y"}}, false, false));
+ assert.eq(staleCollB.findOne().x, "x");
-// Make sure we unsuccessfully update with old info
-assert.writeError(staleCollB.update({c: "c"}, {$set: {x: "y"}}, false, false));
-assert.eq(staleCollB.findOne().x, "x");
+ // Change the collection sharding state
+ coll.drop();
+ coll.ensureIndex({e: 1});
+ // Deletes need to be across two shards to trigger an error - this is probably an exceptional
+ // case
+ assert.commandWorked(admin.runCommand({movePrimary: coll.getDB() + "", to: "shard0000"}));
+ assert.commandWorked(admin.runCommand({shardCollection: coll + "", key: {e: 1}}));
+ assert.commandWorked(admin.runCommand({split: coll + "", middle: {e: 0}}));
+ assert.commandWorked(admin.runCommand({moveChunk: coll + "", find: {e: 0}, to: "shard0001"}));
-// Change the collection sharding state
-coll.drop();
-coll.ensureIndex({e: 1});
-// Deletes need to be across two shards to trigger an error - this is probably an exceptional case
-printjson(admin.runCommand({movePrimary: coll.getDB() + "", to: "shard0000"}));
-printjson(admin.runCommand({shardCollection: coll + "", key: {e: 1}}));
-printjson(admin.runCommand({split: coll + "", middle: {e: 0}}));
-printjson(admin.runCommand({moveChunk: coll + "", find: {e: 0}, to: "shard0001"}));
+ // Make sure we can successfully remove, even though we have stale state
+ assert.writeOK(coll.insert({e: "e"}));
-// Make sure we can successfully remove, even though we have stale state
-assert.writeOK(coll.insert({e: "e"}));
+ assert.writeOK(staleCollA.remove({e: "e"}, true));
+ assert.eq(null, staleCollA.findOne());
-assert.writeOK(staleCollA.remove({e: "e"}, true));
-assert.eq(null, staleCollA.findOne());
+ // Make sure we unsuccessfully remove with old info
+ assert.writeError(staleCollB.remove({d: "d"}, true));
-// Make sure we unsuccessfully remove with old info
-assert.writeError(staleCollB.remove({d: "d"}, true));
-
-st.stop();
+ st.stop();
+})();
diff --git a/src/mongo/dbtests/chunktests.cpp b/src/mongo/dbtests/chunktests.cpp
index fb874f8fecd..5fe6f14e659 100644
--- a/src/mongo/dbtests/chunktests.cpp
+++ b/src/mongo/dbtests/chunktests.cpp
@@ -65,7 +65,7 @@ public:
_chunkMap[mySplitPoints[i]] = chunk;
}
- _chunkRanges.reloadAll(_chunkMap);
+ _chunkRangeMap = std::move(_constructRanges(_chunkMap));
}
};
diff --git a/src/mongo/s/catalog/type_chunk.cpp b/src/mongo/s/catalog/type_chunk.cpp
index 5805642803f..c366f1b1514 100644
--- a/src/mongo/s/catalog/type_chunk.cpp
+++ b/src/mongo/s/catalog/type_chunk.cpp
@@ -52,6 +52,51 @@ const BSONField<bool> ChunkType::jumbo("jumbo");
const BSONField<Date_t> ChunkType::DEPRECATED_lastmod("lastmod");
const BSONField<OID> ChunkType::DEPRECATED_epoch("lastmodEpoch");
+namespace {
+
+const char kMinKey[] = "min";
+const char kMaxKey[] = "max";
+
+} // namespace
+
+ChunkRange::ChunkRange(BSONObj minKey, BSONObj maxKey)
+ : _minKey(std::move(minKey)), _maxKey(std::move(maxKey)) {}
+
+StatusWith<ChunkRange> ChunkRange::fromBSON(const BSONObj& obj) {
+ BSONElement minKey;
+ {
+ Status minKeyStatus = bsonExtractTypedField(obj, kMinKey, Object, &minKey);
+ if (!minKeyStatus.isOK()) {
+ return {minKeyStatus.code(),
+ str::stream() << "Invalid min key due to " << minKeyStatus.reason()};
+ }
+ }
+
+ BSONElement maxKey;
+ {
+ Status maxKeyStatus = bsonExtractTypedField(obj, kMaxKey, Object, &maxKey);
+ if (!maxKeyStatus.isOK()) {
+ return {maxKeyStatus.code(),
+ str::stream() << "Invalid max key due to " << maxKeyStatus.reason()};
+ }
+ }
+
+ return ChunkRange(minKey.Obj().getOwned(), maxKey.Obj().getOwned());
+}
+
+bool ChunkRange::containsKey(const BSONObj& key) const {
+ return _minKey.woCompare(key) <= 0 && key.woCompare(_maxKey) < 0;
+}
+
+void ChunkRange::append(BSONObjBuilder* builder) const {
+ builder->append(kMinKey, _minKey);
+ builder->append(kMaxKey, _maxKey);
+}
+
+std::string ChunkRange::toString() const {
+ return str::stream() << "[" << _minKey << ", " << _maxKey << ")";
+}
+
StatusWith<ChunkType> ChunkType::fromBSON(const BSONObj& source) {
ChunkType chunk;
@@ -72,19 +117,13 @@ StatusWith<ChunkType> ChunkType::fromBSON(const BSONObj& source) {
}
{
- BSONElement chunkMinElement;
- Status status = bsonExtractTypedField(source, min.name(), Object, &chunkMinElement);
- if (!status.isOK())
- return status;
- chunk._min = chunkMinElement.Obj().getOwned();
- }
+ auto chunkRangeStatus = ChunkRange::fromBSON(source);
+ if (!chunkRangeStatus.isOK())
+ return chunkRangeStatus.getStatus();
- {
- BSONElement chunkMaxElement;
- Status status = bsonExtractTypedField(source, max.name(), Object, &chunkMaxElement);
- if (!status.isOK())
- return status;
- chunk._max = chunkMaxElement.Obj().getOwned();
+ const auto chunkRange = std::move(chunkRangeStatus.getValue());
+ chunk._min = chunkRange.getMin().getOwned();
+ chunk._max = chunkRange.getMax().getOwned();
}
{
diff --git a/src/mongo/s/catalog/type_chunk.h b/src/mongo/s/catalog/type_chunk.h
index a13cda9aed9..fbf8e31e654 100644
--- a/src/mongo/s/catalog/type_chunk.h
+++ b/src/mongo/s/catalog/type_chunk.h
@@ -31,17 +31,55 @@
#include <boost/optional.hpp>
#include <string>
-#include "mongo/db/jsobj.h"
+#include "mongo/bson/bsonobj.h"
#include "mongo/s/chunk_version.h"
namespace mongo {
-class BSONObj;
+class BSONObjBuilder;
class Status;
template <typename T>
class StatusWith;
/**
+ * Contains the minimum representation of a chunk - its bounds in the format [min, max) along with
+ * utilities for parsing and persistence.
+ */
+class ChunkRange {
+public:
+ ChunkRange(BSONObj minKey, BSONObj maxKey);
+
+ /**
+ * Parses a chunk range using the format { min: <min bound>, max: <max bound> }.
+ */
+ static StatusWith<ChunkRange> fromBSON(const BSONObj& obj);
+
+ const BSONObj& getMin() const {
+ return _minKey;
+ }
+
+ const BSONObj& getMax() const {
+ return _maxKey;
+ }
+
+ /**
+ * Checks whether the specified key is within the bounds of this chunk range.
+ */
+ bool containsKey(const BSONObj& key) const;
+
+ /**
+ * Writes the contents of this chunk range as { min: <min bound>, max: <max bound> }.
+ */
+ void append(BSONObjBuilder* builder) const;
+
+ std::string toString() const;
+
+private:
+ BSONObj _minKey;
+ BSONObj _maxKey;
+};
+
+/**
* This class represents the layout and contents of documents contained in the
* config.chunks collection. All manipulation of documents coming from that
* collection should be done with this class.
diff --git a/src/mongo/s/chunk.cpp b/src/mongo/s/chunk.cpp
index b6096e3051d..f3d53346825 100644
--- a/src/mongo/s/chunk.cpp
+++ b/src/mongo/s/chunk.cpp
@@ -39,7 +39,6 @@
#include "mongo/s/balancer/balancer.h"
#include "mongo/s/balancer/balancer_configuration.h"
#include "mongo/s/catalog/catalog_manager.h"
-#include "mongo/s/catalog/type_chunk.h"
#include "mongo/s/catalog/type_collection.h"
#include "mongo/s/chunk_manager.h"
#include "mongo/s/client/shard_registry.h"
@@ -218,8 +217,9 @@ std::vector<BSONObj> Chunk::_determineSplitPoints(OperationContext* txn, bool at
return splitPoints;
}
-StatusWith<boost::optional<std::pair<BSONObj, BSONObj>>> Chunk::split(
- OperationContext* txn, SplitPointMode mode, size_t* resultingSplits) const {
+StatusWith<boost::optional<ChunkRange>> Chunk::split(OperationContext* txn,
+ SplitPointMode mode,
+ size_t* resultingSplits) const {
size_t dummy;
if (resultingSplits == NULL) {
resultingSplits = &dummy;
@@ -368,8 +368,8 @@ bool Chunk::splitIfShould(OperationContext* txn, long dataWritten) {
ChunkType chunkToMove;
chunkToMove.setNS(_manager->getns());
chunkToMove.setShard(getShardId());
- chunkToMove.setMin(suggestedMigrateChunk->first);
- chunkToMove.setMax(suggestedMigrateChunk->second);
+ chunkToMove.setMin(suggestedMigrateChunk->getMin());
+ chunkToMove.setMax(suggestedMigrateChunk->getMax());
msgassertedNoTraceWithStatus(
10412, Balancer::get(txn)->rebalanceSingleChunk(txn, chunkToMove));
diff --git a/src/mongo/s/chunk.h b/src/mongo/s/chunk.h
index 28bb1e519cc..9eeabf6e2c2 100644
--- a/src/mongo/s/chunk.h
+++ b/src/mongo/s/chunk.h
@@ -30,6 +30,7 @@
#include <boost/optional.hpp>
+#include "mongo/s/catalog/type_chunk.h"
#include "mongo/s/chunk_version.h"
#include "mongo/s/client/shard.h"
@@ -127,9 +128,9 @@ public:
*
* @throws UserException
*/
- StatusWith<boost::optional<std::pair<BSONObj, BSONObj>>> split(OperationContext* txn,
- SplitPointMode mode,
- size_t* resultingSplits) const;
+ StatusWith<boost::optional<ChunkRange>> split(OperationContext* txn,
+ SplitPointMode mode,
+ size_t* resultingSplits) const;
/**
* marks this chunk as a jumbo chunk
diff --git a/src/mongo/s/chunk_manager.cpp b/src/mongo/s/chunk_manager.cpp
index a7e72788a73..00933658656 100644
--- a/src/mongo/s/chunk_manager.cpp
+++ b/src/mongo/s/chunk_manager.cpp
@@ -32,7 +32,7 @@
#include "mongo/s/chunk_manager.h"
-#include <boost/next_prior.hpp>
+#include <iterator>
#include <map>
#include <set>
@@ -48,7 +48,6 @@
#include "mongo/s/balancer/balancer_configuration.h"
#include "mongo/s/catalog/catalog_cache.h"
#include "mongo/s/catalog/catalog_manager.h"
-#include "mongo/s/catalog/type_chunk.h"
#include "mongo/s/catalog/type_collection.h"
#include "mongo/s/chunk.h"
#include "mongo/s/chunk_diff.h"
@@ -164,15 +163,13 @@ ChunkManager::ChunkManager(const string& ns, const ShardKeyPattern& pattern, boo
: _ns(ns),
_keyPattern(pattern.getKeyPattern()),
_unique(unique),
- _sequenceNumber(NextSequenceNumber.addAndFetch(1)),
- _chunkRanges() {}
+ _sequenceNumber(NextSequenceNumber.addAndFetch(1)) {}
ChunkManager::ChunkManager(const CollectionType& coll)
: _ns(coll.getNs().ns()),
_keyPattern(coll.getKeyPattern()),
_unique(coll.getUnique()),
- _sequenceNumber(NextSequenceNumber.addAndFetch(1)),
- _chunkRanges() {
+ _sequenceNumber(NextSequenceNumber.addAndFetch(1)) {
// coll does not have correct version. Use same initial version as _load and createFirstChunks.
_version = ChunkVersion(0, 0, coll.getEpoch());
}
@@ -199,16 +196,11 @@ void ChunkManager::loadExistingRanges(OperationContext* txn, const ChunkManager*
_chunkMap.swap(chunkMap);
_shardIds.swap(shardIds);
_shardVersions.swap(shardVersions);
- _chunkRanges.reloadAll(_chunkMap);
-
+ _chunkRangeMap = std::move(_constructRanges(_chunkMap));
return;
}
}
- if (_chunkMap.size() < 10) {
- _printChunks();
- }
-
warning() << "ChunkManager loaded an invalid config for " << _ns << ", trying again";
sleepmillis(10 * (3 - tries));
@@ -497,7 +489,7 @@ void ChunkManager::getShardIdsForQuery(OperationContext* txn,
// Query validation
if (QueryPlannerCommon::hasNode(cq->root(), MatchExpression::GEO_NEAR)) {
- uassert(13501, "use geoNear command rather than $near query", false);
+ uasserted(13501, "use geoNear command rather than $near query");
}
// Fast path for targeting equalities on the shard key.
@@ -531,35 +523,36 @@ void ChunkManager::getShardIdsForQuery(OperationContext* txn,
break;
}
- // SERVER-4914 Some clients of getShardIdsForQuery() assume at least one shard will be
- // returned. For now, we satisfy that assumption by adding a shard with no matches rather
- // than return an empty set of shards.
+ // SERVER-4914 Some clients of getShardIdsForQuery() assume at least one shard will be returned.
+ // For now, we satisfy that assumption by adding a shard with no matches rather than returning
+ // an empty set of shards.
if (shardIds->empty()) {
- massert(16068, "no chunk ranges available", !_chunkRanges.ranges().empty());
- shardIds->insert(_chunkRanges.ranges().begin()->second->getShardId());
+ shardIds->insert(_chunkRangeMap.begin()->second.getShardId());
}
}
void ChunkManager::getShardIdsForRange(set<ShardId>& shardIds,
const BSONObj& min,
const BSONObj& max) const {
- using ChunkRangeMap = ChunkRangeManager::ChunkRangeMap;
- ChunkRangeMap::const_iterator it = _chunkRanges.upper_bound(min);
- ChunkRangeMap::const_iterator end = _chunkRanges.upper_bound(max);
+ auto it = _chunkRangeMap.upper_bound(min);
+ auto end = _chunkRangeMap.upper_bound(max);
- massert(13507,
- str::stream() << "no chunks found between bounds " << min << " and " << max,
- it != _chunkRanges.ranges().end());
+ // The chunk range map must always cover the entire key space
+ invariant(it != _chunkRangeMap.end());
- if (end != _chunkRanges.ranges().end())
+ // We need to include the last chunk
+ if (end != _chunkRangeMap.cend()) {
++end;
+ }
for (; it != end; ++it) {
- shardIds.insert(it->second->getShardId());
+ shardIds.insert(it->second.getShardId());
- // once we know we need to visit all shards no need to keep looping
- if (shardIds.size() == _shardIds.size())
+ // No need to iterate through the rest of the ranges, because we already know we need to use
+ // all shards.
+ if (shardIds.size() == _shardIds.size()) {
break;
+ }
}
}
@@ -708,118 +701,43 @@ string ChunkManager::toString() const {
return sb.str();
}
-ChunkManager::ChunkRange::ChunkRange(ChunkMap::const_iterator begin,
- const ChunkMap::const_iterator end)
- : _manager(begin->second->getManager()),
- _shardId(begin->second->getShardId()),
- _min(begin->second->getMin()),
- _max(boost::prior(end)->second->getMax()) {
- invariant(begin != end);
+ChunkManager::ChunkRangeMap ChunkManager::_constructRanges(const ChunkMap& chunkMap) {
+ ChunkRangeMap chunkRangeMap;
- DEV while (begin != end) {
- dassert(begin->second->getManager() == _manager);
- dassert(begin->second->getShardId() == _shardId);
- ++begin;
- }
-}
-
-ChunkManager::ChunkRange::ChunkRange(const ChunkRange& min, const ChunkRange& max)
- : _manager(min.getManager()),
- _shardId(min.getShardId()),
- _min(min.getMin()),
- _max(max.getMax()) {
- invariant(min.getShardId() == max.getShardId());
- invariant(min.getManager() == max.getManager());
- invariant(min.getMax() == max.getMin());
-}
-
-string ChunkManager::ChunkRange::toString() const {
- StringBuilder sb;
- sb << "ChunkRange(min=" << _min << ", max=" << _max << ", shard=" << _shardId << ")";
-
- return sb.str();
-}
-
-bool ChunkManager::ChunkRange::containsKey(const BSONObj& shardKey) const {
- return getMin().woCompare(shardKey) <= 0 && shardKey.woCompare(getMax()) < 0;
-}
-
-void ChunkManager::ChunkRangeManager::_assertValid() const {
- if (_ranges.empty())
- return;
-
- try {
- // No Nulls
- for (ChunkRangeMap::const_iterator it = _ranges.begin(), end = _ranges.end(); it != end;
- ++it) {
- verify(it->second);
- }
-
- // Check endpoints
- verify(allOfType(MinKey, _ranges.begin()->second->getMin()));
- verify(allOfType(MaxKey, boost::prior(_ranges.end())->second->getMax()));
-
- // Make sure there are no gaps or overlaps
- for (ChunkRangeMap::const_iterator it = boost::next(_ranges.begin()), end = _ranges.end();
- it != end;
- ++it) {
- ChunkRangeMap::const_iterator last = boost::prior(it);
- verify(it->second->getMin() == last->second->getMax());
- }
-
- // Check Map keys
- for (ChunkRangeMap::const_iterator it = _ranges.begin(), end = _ranges.end(); it != end;
- ++it) {
- verify(it->first == it->second->getMax());
- }
-
- // Make sure we match the original chunks
- const ChunkMap chunks = _ranges.begin()->second->getManager()->getChunkMap();
- for (ChunkMap::const_iterator i = chunks.begin(); i != chunks.end(); ++i) {
- const ChunkPtr chunk = i->second;
-
- ChunkRangeMap::const_iterator min = _ranges.upper_bound(chunk->getMin());
- ChunkRangeMap::const_iterator max = _ranges.lower_bound(chunk->getMax());
-
- verify(min != _ranges.end());
- verify(max != _ranges.end());
- verify(min == max);
- verify(min->second->getShardId() == chunk->getShardId());
- verify(min->second->containsKey(chunk->getMin()));
- verify(min->second->containsKey(chunk->getMax()) ||
- (min->second->getMax() == chunk->getMax()));
- }
-
- } catch (...) {
- error() << "\t invalid ChunkRangeMap! printing ranges:";
-
- for (ChunkRangeMap::const_iterator it = _ranges.begin(), end = _ranges.end(); it != end;
- ++it) {
- log() << it->first << ": " << it->second->toString();
+ if (chunkMap.empty()) {
+ return chunkRangeMap;
+ }
+
+ ChunkMap::const_iterator current = chunkMap.cbegin();
+
+ while (current != chunkMap.cend()) {
+ const auto rangeFirst = current;
+ current = std::find_if(current,
+ chunkMap.cend(),
+ [&rangeFirst](const ChunkMap::value_type& chunkMapEntry) {
+ return chunkMapEntry.second->getShardId() !=
+ rangeFirst->second->getShardId();
+ });
+ const auto rangeLast = std::prev(current);
+
+ const BSONObj rangeMin = rangeFirst->second->getMin();
+ const BSONObj rangeMax = rangeLast->second->getMax();
+
+ auto insertResult = chunkRangeMap.insert(std::make_pair(
+ rangeMax, ShardAndChunkRange(rangeMin, rangeMax, rangeFirst->second->getShardId())));
+ invariant(insertResult.second);
+ if (insertResult.first != chunkRangeMap.begin()) {
+ // Make sure there are no gaps in the ranges
+ insertResult.first--;
+ invariant(insertResult.first->first == rangeMin);
}
-
- throw;
}
-}
-void ChunkManager::ChunkRangeManager::reloadAll(const ChunkMap& chunks) {
- _ranges.clear();
- _insertRange(chunks.begin(), chunks.end());
+ invariant(!chunkRangeMap.empty());
+ invariant(allOfType(MinKey, chunkRangeMap.begin()->second.getMin()));
+ invariant(allOfType(MaxKey, chunkRangeMap.rbegin()->first));
- DEV _assertValid();
-}
-
-void ChunkManager::ChunkRangeManager::_insertRange(ChunkMap::const_iterator begin,
- const ChunkMap::const_iterator end) {
- while (begin != end) {
- ChunkMap::const_iterator first = begin;
- ShardId shardId = first->second->getShardId();
- while (begin != end && (begin->second->getShardId() == shardId))
- ++begin;
-
- shared_ptr<ChunkRange> cr(new ChunkRange(first, begin));
- _ranges[cr->getMax()] = cr;
- }
+ return chunkRangeMap;
}
uint64_t ChunkManager::getCurrentDesiredChunkSize() const {
diff --git a/src/mongo/s/chunk_manager.h b/src/mongo/s/chunk_manager.h
index 28782148c9d..49009d8abb6 100644
--- a/src/mongo/s/chunk_manager.h
+++ b/src/mongo/s/chunk_manager.h
@@ -34,6 +34,7 @@
#include <vector>
#include "mongo/db/repl/optime.h"
+#include "mongo/s/catalog/type_chunk.h"
#include "mongo/s/chunk.h"
#include "mongo/s/chunk_version.h"
#include "mongo/s/client/shard.h"
@@ -183,85 +184,52 @@ public:
repl::OpTime getConfigOpTime() const;
private:
- class ChunkRange {
+ /**
+ * Represents a range of chunk keys [getMin(), getMax()) and the id of the shard on which they
+ * reside according to the metadata.
+ */
+ class ShardAndChunkRange {
public:
- ChunkRange(ChunkMap::const_iterator begin, const ChunkMap::const_iterator end);
-
- // Merge min and max (must be adjacent ranges)
- ChunkRange(const ChunkRange& min, const ChunkRange& max);
-
- const ChunkManager* getManager() const {
- return _manager;
- }
-
- ShardId getShardId() const {
- return _shardId;
- }
+ ShardAndChunkRange(const BSONObj& min, const BSONObj& max, ShardId inShardId)
+ : _range(min, max), _shardId(std::move(inShardId)) {}
const BSONObj& getMin() const {
- return _min;
+ return _range.getMin();
}
const BSONObj& getMax() const {
- return _max;
- }
-
- // clones of Chunk methods
- // Returns true if this ChunkRange contains the given shard key, and false otherwise
- //
- // Note: this function takes an extracted *key*, not an original document
- // (the point may be computed by, say, hashing a given field or projecting
- // to a subset of fields).
- bool containsKey(const BSONObj& shardKey) const;
-
- std::string toString() const;
-
- private:
- const ChunkManager* _manager;
- const ShardId _shardId;
- const BSONObj _min;
- const BSONObj _max;
- };
-
- class ChunkRangeManager {
- public:
- using ChunkRangeMap = std::map<BSONObj, std::shared_ptr<ChunkRange>, BSONObjCmp>;
-
- const ChunkRangeMap& ranges() const {
- return _ranges;
- }
-
- void clear() {
- _ranges.clear();
+ return _range.getMax();
}
- void reloadAll(const ChunkMap& chunks);
-
- ChunkRangeMap::const_iterator upper_bound(const BSONObj& o) const {
- return _ranges.upper_bound(o);
- }
-
- ChunkRangeMap::const_iterator lower_bound(const BSONObj& o) const {
- return _ranges.lower_bound(o);
+ const ShardId& getShardId() const {
+ return _shardId;
}
private:
- // assumes nothing in this range exists in _ranges
- void _insertRange(ChunkMap::const_iterator begin, const ChunkMap::const_iterator end);
-
- // Slow operation -- wrap with DEV
- void _assertValid() const;
-
- ChunkRangeMap _ranges;
+ ChunkRange _range;
+ ShardId _shardId;
};
- // returns true if load was consistent
+ // Contains a compressed map of what range of keys resides on which shard. The index is the max
+ // key of the respective range and the union of all ranges in a such constructed map must cover
+ // the complete space from [MinKey, MaxKey).
+ using ChunkRangeMap = std::map<BSONObj, ShardAndChunkRange, BSONObjCmp>;
+
+ /**
+ * If load was successful, returns true and it is guaranteed that the _chunkMap and
+ * _chunkRangeMap are consistent with each other. If false is returned, it is not safe to use
+ * the chunk manager anymore.
+ */
bool _load(OperationContext* txn,
ChunkMap& chunks,
std::set<ShardId>& shardIds,
ShardVersionMap* shardVersions,
const ChunkManager* oldManager);
+ /**
+ * Merges consecutive chunks, which reside on the same shard into a single range.
+ */
+ static ChunkRangeMap _constructRanges(const ChunkMap& chunkMap);
// All members should be const for thread-safety
const std::string _ns;
@@ -274,7 +242,7 @@ private:
const unsigned long long _sequenceNumber;
ChunkMap _chunkMap;
- ChunkRangeManager _chunkRanges;
+ ChunkRangeMap _chunkRangeMap;
std::set<ShardId> _shardIds;
diff --git a/src/mongo/s/shard_util.cpp b/src/mongo/s/shard_util.cpp
index aa15dc03722..7de40831818 100644
--- a/src/mongo/s/shard_util.cpp
+++ b/src/mongo/s/shard_util.cpp
@@ -51,35 +51,6 @@ const char kMinKey[] = "min";
const char kMaxKey[] = "max";
const char kShouldMigrate[] = "shouldMigrate";
-/**
- * Extracts the bounds of a chunk from a BSON object having the following format:
- * { min: <min key>, max: <max key> }
- *
- * Returns a failed status if the format cannot be matched.
- */
-StatusWith<std::pair<BSONObj, BSONObj>> extractChunkBounds(const BSONObj& obj) {
- BSONElement minKey;
- {
- Status minKeyStatus = bsonExtractTypedField(obj, kMinKey, Object, &minKey);
- if (!minKeyStatus.isOK()) {
- return {minKeyStatus.code(),
- str::stream() << "Invalid min key due to " << minKeyStatus.toString()};
- }
- }
-
- BSONElement maxKey;
- {
- Status maxKeyStatus = bsonExtractTypedField(obj, kMaxKey, Object, &maxKey);
- if (!maxKeyStatus.isOK()) {
- return {maxKeyStatus.code(),
- str::stream() << "Invalid max key due to " << maxKeyStatus.toString()};
- }
- }
-
- return std::pair<BSONObj, BSONObj>(
- std::make_pair(minKey.Obj().getOwned(), maxKey.Obj().getOwned()));
-}
-
} // namespace
StatusWith<long long> retrieveTotalShardSize(OperationContext* txn, const ShardId& shardId) {
@@ -196,7 +167,7 @@ StatusWith<std::vector<BSONObj>> selectChunkSplitPoints(OperationContext* txn,
return std::move(splitPoints);
}
-StatusWith<boost::optional<std::pair<BSONObj, BSONObj>>> splitChunkAtMultiplePoints(
+StatusWith<boost::optional<ChunkRange>> splitChunkAtMultiplePoints(
OperationContext* txn,
const ShardId& shardId,
const NamespaceString& nss,
@@ -258,13 +229,12 @@ StatusWith<boost::optional<std::pair<BSONObj, BSONObj>>> splitChunkAtMultiplePoi
BSONElement shouldMigrateElement;
status = bsonExtractTypedField(cmdResponse, kShouldMigrate, Object, &shouldMigrateElement);
if (status.isOK()) {
- auto chunkBoundsStatus = extractChunkBounds(shouldMigrateElement.embeddedObject());
- if (!chunkBoundsStatus.isOK()) {
- return chunkBoundsStatus.getStatus();
+ auto chunkRangeStatus = ChunkRange::fromBSON(shouldMigrateElement.embeddedObject());
+ if (!chunkRangeStatus.isOK()) {
+ return chunkRangeStatus.getStatus();
}
- return boost::optional<std::pair<BSONObj, BSONObj>>(
- std::move(chunkBoundsStatus.getValue()));
+ return boost::optional<ChunkRange>(std::move(chunkRangeStatus.getValue()));
} else if (status != ErrorCodes::NoSuchKey) {
warning()
<< "Chunk migration will be skipped because splitChunk returned invalid response: "
@@ -272,7 +242,7 @@ StatusWith<boost::optional<std::pair<BSONObj, BSONObj>>> splitChunkAtMultiplePoi
<< causedBy(status);
}
- return boost::optional<std::pair<BSONObj, BSONObj>>();
+ return boost::optional<ChunkRange>();
}
} // namespace shardutil
diff --git a/src/mongo/s/shard_util.h b/src/mongo/s/shard_util.h
index 98ea7bc2f30..2edc7025db4 100644
--- a/src/mongo/s/shard_util.h
+++ b/src/mongo/s/shard_util.h
@@ -32,6 +32,7 @@
#include <string>
#include <vector>
+#include "mongo/s/catalog/type_chunk.h"
#include "mongo/s/chunk_version.h"
#include "mongo/s/client/shard.h"
@@ -103,7 +104,7 @@ StatusWith<std::vector<BSONObj>> selectChunkSplitPoints(OperationContext* txn,
* minKey/maxKey Bounds of the chunk to be split.
* splitPoints The set of points at which the chunk should be split.
*/
-StatusWith<boost::optional<std::pair<BSONObj, BSONObj>>> splitChunkAtMultiplePoints(
+StatusWith<boost::optional<ChunkRange>> splitChunkAtMultiplePoints(
OperationContext* txn,
const ShardId& shardId,
const NamespaceString& nss,