org.apache.cassandra.dht.Range Maven / Gradle / Ivy
Go to download
Show more of this group Show more artifacts with this name
Show all versions of cassandra-all Show documentation
Show all versions of cassandra-all Show documentation
The Apache Cassandra Project develops a highly scalable second-generation distributed database, bringing together Dynamo's fully distributed design and Bigtable's ColumnFamily-based data model.
/*
* 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.cassandra.dht;
import java.io.Serializable;
import java.util.*;
import org.apache.commons.lang3.ObjectUtils;
import org.apache.cassandra.db.ColumnFamilyStore;
import org.apache.cassandra.db.RowPosition;
import org.apache.cassandra.service.StorageService;
import org.apache.cassandra.utils.Pair;
/**
* A representation of the range that a node is responsible for on the DHT ring.
*
* A Range is responsible for the tokens between (left, right].
*
* Used by the partitioner and by map/reduce by-token range scans.
*/
public class Range> extends AbstractBounds implements Comparable>, Serializable
{
public static final long serialVersionUID = 1L;
public Range(T left, T right)
{
this(left, right, StorageService.getPartitioner());
}
public Range(T left, T right, IPartitioner partitioner)
{
super(left, right, partitioner);
}
public static > boolean contains(T left, T right, T point)
{
if (isWrapAround(left, right))
{
/*
* We are wrapping around, so the interval is (a,b] where a >= b,
* then we have 3 cases which hold for any given token k:
* (1) a < k -- return true
* (2) k <= b -- return true
* (3) b < k <= a -- return false
*/
if (point.compareTo(left) > 0)
return true;
else
return right.compareTo(point) >= 0;
}
else
{
/*
* This is the range (a, b] where a < b.
*/
return point.compareTo(left) > 0 && right.compareTo(point) >= 0;
}
}
public boolean contains(Range that)
{
if (this.left.equals(this.right))
{
// full ring always contains all other ranges
return true;
}
boolean thiswraps = isWrapAround(left, right);
boolean thatwraps = isWrapAround(that.left, that.right);
if (thiswraps == thatwraps)
{
return left.compareTo(that.left) <= 0 && that.right.compareTo(right) <= 0;
}
else if (thiswraps)
{
// wrapping might contain non-wrapping
// that is contained if both its tokens are in one of our wrap segments
return left.compareTo(that.left) <= 0 || that.right.compareTo(right) <= 0;
}
else
{
// (thatwraps)
// non-wrapping cannot contain wrapping
return false;
}
}
/**
* Helps determine if a given point on the DHT ring is contained
* in the range in question.
* @param point point in question
* @return true if the point contains within the range else false.
*/
public boolean contains(T point)
{
return contains(left, right, point);
}
/**
* @param that range to check for intersection
* @return true if the given range intersects with this range.
*/
public boolean intersects(Range that)
{
return intersectionWith(that).size() > 0;
}
public boolean intersects(AbstractBounds that)
{
// implemented for cleanup compaction membership test, so only Range + Bounds are supported for now
if (that instanceof Range)
return intersects((Range) that);
if (that instanceof Bounds)
return intersects((Bounds) that);
throw new UnsupportedOperationException("Intersection is only supported for Bounds and Range objects; found " + that.getClass());
}
/**
* @param that range to check for intersection
* @return true if the given range intersects with this range.
*/
public boolean intersects(Bounds that)
{
// Same punishment than in Bounds.contains(), we must be carefull if that.left == that.right as
// as new Range(that.left, that.right) will then cover the full ring which is not what we
// want.
return contains(that.left) || (!that.left.equals(that.right) && intersects(new Range(that.left, that.right)));
}
@SafeVarargs
public static > Set> rangeSet(Range ... ranges)
{
return Collections.unmodifiableSet(new HashSet>(Arrays.asList(ranges)));
}
public static > Set> rangeSet(Range range)
{
return Collections.singleton(range);
}
/**
* @param that
* @return the intersection of the two Ranges. this can be two disjoint Ranges if one is wrapping and one is not.
* say you have nodes G and M, with query range (D,T]; the intersection is (M-T] and (D-G].
* If there is no intersection, an empty list is returned.
*/
public Set> intersectionWith(Range that)
{
if (that.contains(this))
return rangeSet(this);
if (this.contains(that))
return rangeSet(that);
boolean thiswraps = isWrapAround(left, right);
boolean thatwraps = isWrapAround(that.left, that.right);
if (!thiswraps && !thatwraps)
{
// neither wraps. the straightforward case.
if (!(left.compareTo(that.right) < 0 && that.left.compareTo(right) < 0))
return Collections.emptySet();
return rangeSet(new Range(ObjectUtils.max(this.left, that.left),
ObjectUtils.min(this.right, that.right),
partitioner));
}
if (thiswraps && thatwraps)
{
// if the starts are the same, one contains the other, which we have already ruled out.
assert !this.left.equals(that.left);
// two wrapping ranges always intersect.
// since we have already determined that neither this nor that contains the other, we have 2 cases,
// and mirror images of those case.
// (1) both of that's (1, 2] endpoints lie in this's (A, B] right segment:
// ---------B--------A--1----2------>
// (2) only that's start endpoint lies in this's right segment:
// ---------B----1---A-------2------>
// or, we have the same cases on the left segement, which we can handle by swapping this and that.
return this.left.compareTo(that.left) < 0
? intersectionBothWrapping(this, that)
: intersectionBothWrapping(that, this);
}
if (thiswraps && !thatwraps)
return intersectionOneWrapping(this, that);
assert (!thiswraps && thatwraps);
return intersectionOneWrapping(that, this);
}
private static > Set> intersectionBothWrapping(Range first, Range that)
{
Set> intersection = new HashSet>(2);
if (that.right.compareTo(first.left) > 0)
intersection.add(new Range(first.left, that.right, first.partitioner));
intersection.add(new Range(that.left, first.right, first.partitioner));
return Collections.unmodifiableSet(intersection);
}
private static > Set> intersectionOneWrapping(Range wrapping, Range other)
{
Set> intersection = new HashSet>(2);
if (other.contains(wrapping.right))
intersection.add(new Range(other.left, wrapping.right, wrapping.partitioner));
// need the extra compareto here because ranges are asymmetrical; wrapping.left _is not_ contained by the wrapping range
if (other.contains(wrapping.left) && wrapping.left.compareTo(other.right) < 0)
intersection.add(new Range(wrapping.left, other.right, wrapping.partitioner));
return Collections.unmodifiableSet(intersection);
}
public Pair, AbstractBounds> split(T position)
{
assert contains(position) || left.equals(position);
// Check if the split would have no effect on the range
if (position.equals(left) || position.equals(right))
return null;
AbstractBounds lb = new Range(left, position, partitioner);
AbstractBounds rb = new Range(position, right, partitioner);
return Pair.create(lb, rb);
}
public boolean inclusiveLeft()
{
return false;
}
public boolean inclusiveRight()
{
return true;
}
public List> unwrap()
{
@SuppressWarnings("unchecked")
T minValue = (T) partitioner.minValue(right.getClass());
if (!isWrapAround() || right.equals(minValue))
return Arrays.asList(this);
List> unwrapped = new ArrayList>(2);
unwrapped.add(new Range(left, minValue, partitioner));
unwrapped.add(new Range(minValue, right, partitioner));
return unwrapped;
}
/**
* Tells if the given range is a wrap around.
*/
public static > boolean isWrapAround(T left, T right)
{
return left.compareTo(right) >= 0;
}
public int compareTo(Range rhs)
{
/*
* If the range represented by the "this" pointer
* is a wrap around then it is the smaller one.
*/
if ( isWrapAround(left, right) )
return -1;
if ( isWrapAround(rhs.left, rhs.right) )
return 1;
return right.compareTo(rhs.right);
}
/**
* Subtracts a portion of this range.
* @param contained The range to subtract from this. It must be totally
* contained by this range.
* @return An ArrayList of the Ranges left after subtracting contained
* from this.
*/
private ArrayList> subtractContained(Range contained)
{
ArrayList> difference = new ArrayList>(2);
if (!left.equals(contained.left))
difference.add(new Range(left, contained.left, partitioner));
if (!right.equals(contained.right))
difference.add(new Range(contained.right, right, partitioner));
return difference;
}
public Set> subtract(Range rhs)
{
return rhs.differenceToFetch(this);
}
public Set> subtractAll(Collection> ranges)
{
Set> result = new HashSet<>();
result.add(this);
for(Range range : ranges)
{
result = substractAllFromToken(result, range);
}
return result;
}
private static > Set> substractAllFromToken(Set> ranges, Range subtract)
{
Set> result = new HashSet<>();
for(Range range : ranges)
{
result.addAll(range.subtract(subtract));
}
return result;
}
/**
* Calculate set of the difference ranges of given two ranges
* (as current (A, B] and rhs is (C, D])
* which node will need to fetch when moving to a given new token
*
* @param rhs range to calculate difference
* @return set of difference ranges
*/
public Set> differenceToFetch(Range rhs)
{
Set> result;
Set> intersectionSet = this.intersectionWith(rhs);
if (intersectionSet.isEmpty())
{
result = new HashSet>();
result.add(rhs);
}
else
{
@SuppressWarnings("unchecked")
Range[] intersections = (Range[]) new Range[intersectionSet.size()];
intersectionSet.toArray(intersections);
if (intersections.length == 1)
{
result = new HashSet>(rhs.subtractContained(intersections[0]));
}
else
{
// intersections.length must be 2
Range first = intersections[0];
Range second = intersections[1];
ArrayList> temp = rhs.subtractContained(first);
// Because there are two intersections, subtracting only one of them
// will yield a single Range.
Range single = temp.get(0);
result = new HashSet>(single.subtractContained(second));
}
}
return result;
}
public static > boolean isInRanges(T token, Iterable> ranges)
{
assert ranges != null;
for (Range range : ranges)
{
if (range.contains(token))
{
return true;
}
}
return false;
}
@Override
public boolean equals(Object o)
{
if (!(o instanceof Range))
return false;
Range> rhs = (Range>)o;
return left.equals(rhs.left) && right.equals(rhs.right);
}
@Override
public String toString()
{
return "(" + left + "," + right + "]";
}
protected String getOpeningString()
{
return "(";
}
protected String getClosingString()
{
return "]";
}
public List asList()
{
ArrayList ret = new ArrayList(2);
ret.add(left.toString());
ret.add(right.toString());
return ret;
}
public boolean isWrapAround()
{
return isWrapAround(left, right);
}
/**
* @return A copy of the given list of with all ranges unwrapped, sorted by left bound and with overlapping bounds merged.
*/
public static > List> normalize(Collection> ranges)
{
// unwrap all
List> output = new ArrayList>(ranges.size());
for (Range range : ranges)
output.addAll(range.unwrap());
// sort by left
Collections.sort(output, new Comparator>()
{
public int compare(Range b1, Range b2)
{
return b1.left.compareTo(b2.left);
}
});
// deoverlap
return deoverlap(output);
}
/**
* Given a list of unwrapped ranges sorted by left position, return an
* equivalent list of ranges but with no overlapping ranges.
*/
private static > List> deoverlap(List> ranges)
{
if (ranges.isEmpty())
return ranges;
List> output = new ArrayList>();
Iterator> iter = ranges.iterator();
Range current = iter.next();
@SuppressWarnings("unchecked")
T min = (T) current.partitioner.minValue(current.left.getClass());
while (iter.hasNext())
{
// If current goes to the end of the ring, we're done
if (current.right.equals(min))
{
// If one range is the full range, we return only that
if (current.left.equals(min))
return Collections.>singletonList(current);
output.add(new Range(current.left, min));
return output;
}
Range next = iter.next();
// if next left is equal to current right, we do not intersect per se, but replacing (A, B] and (B, C] by (A, C] is
// legit, and since this avoid special casing and will result in more "optimal" ranges, we do the transformation
if (next.left.compareTo(current.right) <= 0)
{
// We do overlap
// (we've handled current.right.equals(min) already)
if (next.right.equals(min) || current.right.compareTo(next.right) < 0)
current = new Range(current.left, next.right);
}
else
{
output.add(current);
current = next;
}
}
output.add(current);
return output;
}
/**
* Compute a range of keys corresponding to a given range of token.
*/
public static Range makeRowRange(Token left, Token right, IPartitioner partitioner)
{
return new Range(left.maxKeyBound(partitioner), right.maxKeyBound(partitioner), partitioner);
}
@SuppressWarnings("unchecked")
public AbstractBounds toRowBounds()
{
return (left instanceof Token) ? makeRowRange((Token)left, (Token)right, partitioner) : (Range)this;
}
@SuppressWarnings("unchecked")
public AbstractBounds toTokenBounds()
{
return (left instanceof RowPosition) ? new Range(((RowPosition)left).getToken(), ((RowPosition)right).getToken(), partitioner) : (Range)this;
}
public AbstractBounds withNewRight(T newRight)
{
return new Range(left, newRight);
}
/**
* Helper class to check if a token is contained within a given collection of ranges
*/
public static class OrderedRangeContainmentChecker
{
private final Iterator> normalizedRangesIterator;
private Token lastToken = null;
private Range currentRange;
public OrderedRangeContainmentChecker(Collection> ranges)
{
normalizedRangesIterator = normalize(ranges).iterator();
assert normalizedRangesIterator.hasNext();
currentRange = normalizedRangesIterator.next();
}
/**
* Returns true if the ranges given in the constructor contains the token, false otherwise.
*
* The tokens passed to this method must be in increasing order
*
* @param t token to check, must be larger than or equal to the last token passed
* @return true if the token is contained within the ranges given to the constructor.
*/
public boolean contains(Token t)
{
assert lastToken == null || lastToken.compareTo(t) <= 0;
lastToken = t;
while (true)
{
if (t.compareTo(currentRange.left) <= 0)
return false;
else if (t.compareTo(currentRange.right) <= 0 || currentRange.right.compareTo(currentRange.left) <= 0)
return true;
if (!normalizedRangesIterator.hasNext())
return false;
currentRange = normalizedRangesIterator.next();
}
}
}
}