hlc

package
v0.0.2 Latest Latest
Warning

This package is not in the latest version of its module.

Go to latest
Published: Mar 20, 2022 License: Apache-2.0 Imports: 19 Imported by: 0

Documentation

Overview

Package hlc implements the Hybrid Logical Clock outlined in "Logical Physical Clocks and Consistent Snapshots in Globally Distributed Databases", available online at http://www.cse.buffalo.edu/tech-reports/2014-04.pdf.

Each node within a CockroachDB cluster maintains a hybrid logical clock (HLC), which provides timestamps that are a combination of physical and logical time. Physical time is based on a node’s coarsely-synchronized system clock, and logical time is based on Lamport’s clocks.

Hybrid-logical clocks provide a few important properties:

Causality tracking

HLCs provide causality tracking through a combination of a physical and logical (to break ties) component upon each inter-node exchange. Nodes attach HLC timestamps to each message that they send and use HLC timestamps from each message that they receive to update their local clock.

There are currently three channels through which HLC timestamps are passed between nodes in a cluster:

  • Raft (unidirectional): proposers of Raft commands (i.e. leaseholders) attach clock readings to some of these command (e.g. lease transfers, range merges), which are later consumed by followers when commands are applied to their Raft state machine.

    Ref: (roachpb.Lease).Start. Ref: (roachpb.MergeTrigger).FreezeStart.

  • BatchRequest API (bidirectional): clients and servers of the KV BatchRequest API will attach HLC clock readings on requests and responses (successes and errors).

    Ref: (roachpb.Header).Timestamp. Ref: (roachpb.BatchResponse_Header).Now. Ref: (roachpb.Error).Now.

  • DistSQL flows (unidirectional): leaves of a DistSQL flow will pass clock readings back to the root of the flow. Currently, this only takes place on errors, and relates to the "Transaction retry errors" interaction detailed below.

    Ref: (roachpb.Error).Now.

Capturing causal relationships between events on different nodes is critical for enforcing invariants within CockroachDB. What follows is an enumeration of each of the interactions in which causality passing through an HLC is necessary for correctness. It is intended to be exhaustive.

For context, recall CockroachDB's transactional model: we guarantee serializability (transactions appear to be executed in _some_ serial order), and linearizability (transactions appear to execute in real-time order) for transactions that have overlapping read/write sets. Notably, we are not strictly serializable, so transactions with disjoint read/write sets can appear to execute in any order to a concurrent third observer that has overlapping read/write sets with both transactions (deemed a "causal reverse").

The linearizability guarantee is important to note as two sequential (in real time) transactions via two different gateway nodes can be assigned timestamps in reverse order (the second gateway's clock may be behind), but must still see results according to real-time order if they access overlapping keys (e.g. B must see A's write). Also keep in mind that an intent's written timestamp signifies when the intent itself was written, but the final value will be resolved to the transaction's commit timestamp, which may be later than the written timestamp. Since the commit status and timestamp are non-local properties, a range may contain committed values (as unresolved intents) that turn out to exist in the future of the local HLC when the intent gets resolved.

TODO(nvanbenschoten): Update the above on written timestamps after #72121.

  • Cooperative lease transfers (Raft channel). During a cooperative lease transfer from one replica of a range to another, the outgoing leaseholder revokes its lease before its expiration time and consults its clock to determine the start time of the next lease. It then proposes this new lease through Raft (see the raft channel above). Upon application of this Raft entry, the incoming leaseholder forwards its HLC to the start time of the lease, ensuring that its clock is >= the new lease's start time.

    The invariant that a leaseholder's clock is always >= its lease's start time is used in a few places. First, it ensures that the leaseholder's clock always leads the written_timestamp of any value in its keyspace written by a prior leaseholder on its range, which is an important property for the correctness of observed timestamps. Second, it ensures that the leaseholder immediately views itself as the leaseholder. Third, it ensures that if the new leaseholder was to transfer the lease away at some point in the future, this later lease's start time could be pulled from the local clock and be guaranteed to receive an even greater starting timestamp.

    TODO(nvanbenschoten): the written_timestamp concept does not yet exist in code. It will be introduced in the replacement to #72121.

  • Range merges (Raft + BatchRequest channels). During a merge of two ranges, the right-hand side of the merge passes a "frozen timestamp" clock reading from the right-hand side leaseholder, through the merge transaction coordinator, all the way to the left-hand side's leaseholder. This timestamp is captured after the right-hand side has been subsumed and has stopped serving KV traffic. When the left-hand side's leaseholder applies the range merge and officially takes control of the combined range, it forwards its HLC to this frozen timestamp. Like the previous interaction, this one is also necessary to ensure that the leaseholder of the joint range has a clock that leads the written_timestamp of any value in its keyspace, even one written originally on the right-hand side range.

  • Observed timestamps (Raft + BatchRequest channels). During the lifetime of a transaction, its coordinator issues BatchRequests to other nodes in the cluster. Each time a given transaction visits a node for the first time, it captures an observation from the node's HLC. Separately, when a leaseholder on a given node serves a write, it ensures that the node's HLC clock is >= the written_timestamp of the write. This written_timestamp is retained even if an intent is moved to a higher timestamp if it is asynchronously resolved. As a result, these "observed timestamps" captured during the lifetime of a transaction can be used to make a claim about values that could not have been written yet at the time that the transaction first visited the node, and by extension, at the time that the transaction began. This allows the transaction to avoid uncertainty restarts in some circumstances.

    A variant of this same mechanism applies to non-transactional requests that defer their timestamp allocation to the leaseholder of their (single) range. These requests do not collect observed timestamps directly, but they do establish an uncertainty interval immediately upon receipt by their target leaseholder, using a clock reading from the leaseholder's local HLC as the local limit and this clock reading + the cluster's maximum clock skew as the global limit. This limit can be used to make claims about values that could not have been written yet at the time that the non-transaction request first reached the leaseholder node.

    For more, see pkg/kv/kvserver/uncertainty/doc.go.

  • Transaction retry errors (BatchRequest and DistSQL channels). TODO(nvanbenschoten/andreimatei): is this a real case where passing a remote clock signal through the local clock is necessary? The DistSQL channel and its introduction in 72fa944 seem to imply that it is, but I don't really see it, because we don't use the local clock to determine which timestamp to restart the transaction at. Maybe we were just concerned about a transaction restarting at a timestamp above the local clock back then because we had yet to separate the "clock timestamp" domain from the "transaction timestamp" domain.

Strict monotonicity

HLCs, as implemented by CockroachDB, provide strict monotonicity within and across restarts on a single node. Within a continuous process, providing this property is trivial. Across restarts, this property is enforced by waiting out the maximum clock offset upon process startup before serving any requests in ensureClockMonotonicity.

The cluster setting server.clock.persist_upper_bound_interval also provides additional protection here by persisting the wall time of the clock periodically, although this protection is disabled by default.

Strictly monotonic timestamp allocation ensures that two causally dependent transactions originating from the same node are given timestamps that reflect their ordering in real time. However, this itself is not a crucial guarantee, as CockroachDB does not and can not assume that causally dependent operations originate from the same node.

Strictly monotonic timestamp allocation underpins the causality tracking uses detailed above.

Self-stabilization

HLCs provide self-stabilization in the presence of isolated transient clock skew fluctuations. As stated above, a node forwards its HLC upon its receipt of a network message. The effect of this is that given sufficient intra-cluster communication, HLCs across nodes tend to converge and stabilize even if their individual physical clocks diverge. This provides no strong guarantees but can mask clock synchronization errors in practice.

Bounded skew

HLCs within a CockroachDB deployment are configured with a maximum allowable offset between their physical time component and that of other HLCs in the cluster. This is referred to as the MaxOffset. The configuration is static and must be identically configured on all nodes in a cluster, which is enforced by the HeartbeatService.

The notion of a maximum clock skew at all times between all HLCs in a cluster is a foundational assumption used in different parts of the system. What follows is an enumeration of the interactions that assume a bounded clock skew, along with a discussion for each about the consequences of that assumption being broken and the maximum clock skew between two nodes in the cluster exceeding the configured limit.

  • Transaction uncertainty intervals. The single-key linearizability property is satisfied in CockroachDB by tracking an uncertainty interval for each transaction, within which the real-time ordering between two transactions is indeterminate. Upon its creation, a transaction is given a provisional commit timestamp commit_ts from the transaction coordinator’s local HLC and an uncertainty interval of [commit_ts, commit_ts + max_offset].

    When a transaction encounters a value on a key at a timestamp below its provisional commit timestamp, it trivially observes the value during reads and overwrites the value at a higher timestamp during writes. This alone would satisfy single-key linearizability if transactions had access to a perfectly synchronized global clock.

    Without global synchronization, the uncertainty interval is needed because it is possible for a transaction to receive a provisional commit timestamp up to the cluster’s max_offset earlier than a transaction that causally preceded this new transaction in real time. When a transaction encounters a value on a key at a timestamp above its provisional commit timestamp but within its uncertainty interval, it performs an uncertainty restart, moving its provisional commit timestamp above the uncertain value but keeping the upper bound of its uncertainty interval fixed.

    This corresponds to treating all values in a transaction’s uncertainty window as past writes. As a result, the operations on each key performed by transactions take place in an order consistent with the real time ordering of those transactions.

    HAZARD: If the maximum clock offset is exceeded, it is possible for a transaction to serve a stale read that violates single-key linearizability. For example, it is possible for a transaction A to write to a key and commit at a timestamp t1, then for its client to hear about the commit. The client may then initiate a second transaction B on a different gateway that has a slow clock. If this slow clock is more than max_offset from other clocks in the system, it is possible for transaction B's uncertainty interval not to extend up to t1 and therefore for a read of the key that transaction A wrote to be missed. Notably, this is a violation of consistency (linearizability) but not of isolation (serializability) — transaction isolation has no clock dependence.

  • Non-cooperative lease transfers. In the happy case, range leases move from replica to replica using a coordinated handoff. However, in the unhappy case where a leaseholder crashes or otherwise becomes unresponsive, other replicas are able to attempt to acquire a new lease for the range as soon as they observe the old lease expire. In this case, the max_offset plays a role in ensuring that two replicas do not both consider themselves the leaseholder for a range at the same (wallclock) time. This is ensured by designating a "stasis" period equal in size to the max_offset at the end of each lease, immediately before its expiration, as unusable. By preventing a lease from being used within this stasis period, two replicas will never think that they hold a valid lease at the same time, even if the outgoing leaseholder has a slow clock and the incoming leaseholder has a fast clock (within bounds). For more, see LeaseState_UNUSABLE.

    Note however that it is easy to overstate the salient point here if one is not careful. Lease start and end times operate in the MVCC time domain, and any two leases are always guaranteed to cover disjoint intervals of MVCC time. Leases entitle their holder to serve reads at any MVCC time below their expiration and to serve writes at any MVCC time at or after their start time and below their expiration. Additionally, the lease sequence is attached to all writes and checked during Raft application, so a stale leaseholder is unable to perform a write after it has been replaced (in "consensus time"). This combines to mean that even if two replicas believed that they hold the lease for a range at the same time, they can not perform operations that would be incompatible with one another (e.g. two conflicting writes). Again, transaction isolation has no clock dependence.

    HAZARD: If the maximum clock offset is exceeded, it is possible for two replicas to both consider themselves leaseholders at the same time. This can not lead to stale reads for transactional requests, because a transaction with an uncertainty interval that extends past a lease's expiration will not be able to use that lease to perform a read (which is enforced by a stasis period immediately before its expiration). However, because some non-transactional requests receive their timestamp on the server and do not carry an uncertainty interval, they would be susceptible to stale reads during this period. This is equivalent to the hazard for operations that do use uncertainty intervals, but the mechanics differ slightly.

  • "Linearizable" transactions. By default, transactions in CockroachDB provide single-key linearizability and guarantee that as long as clock skew remains below the configured bounds, transactions will not serve stale reads. However, by default, transactions do not provide strict serializability, as they are susceptible to the "causal reverse" anomaly.

    However, CockroachDB does supports a stricter model of consistency through its COCKROACH_EXPERIMENTAL_LINEARIZABLE environment variable. When in "linearizable" mode (also known as "strict serializable" mode), all writing transactions (but not read-only transactions) must wait ("commit-wait") an additional max_offset after committing to ensure that their commit timestamp is below the current HLC clock time of any other node in the system. In doing so, all causally dependent transactions are guaranteed to start with higher timestamps, regardless of the gateway they use. This ensures that all causally dependent transactions commit with higher timestamps, even if their read and writes sets do not conflict with the original transaction's. This prevents the "causal reverse" anomaly which can be observed by a third, concurrent transaction.

    HAZARD: If the maximum clock offset is exceeded, it is possible that even after a transaction commit-waits for the full max_offset, a causally dependent transaction that evaluates on a different gateway node receives and commits with an earlier timestamp. This resuscitates the possibility of the causal reverse anomaly, along with the possibility for stale reads, as detailed above.

    HAZARD: This mode of operation is completely untested.

To reduce the likelihood of stale reads, nodes periodically measure their clock’s offset from other nodes. If any node exceeds the configured maximum offset by more than 80% compared to a majority of other nodes, the node in the minority self-terminates. This best-effort validation is done in (rpc.RemoteClockMonitor).VerifyClockOffset.

Index

Constants

This section is empty.

Variables

View Source
var (
	ErrInvalidLengthLegacyTimestamp        = fmt.Errorf("proto: negative length found during unmarshaling")
	ErrIntOverflowLegacyTimestamp          = fmt.Errorf("proto: integer overflow")
	ErrUnexpectedEndOfGroupLegacyTimestamp = fmt.Errorf("proto: unexpected end of group")
)
View Source
var (
	// MaxTimestamp is the max value allowed for Timestamp.
	MaxTimestamp = Timestamp{WallTime: math.MaxInt64, Logical: math.MaxInt32}
	// MinTimestamp is the min value allowed for Timestamp.
	MinTimestamp = Timestamp{WallTime: 0, Logical: 1}
	// MaxClockTimestamp is the max value allowed for ClockTimestamp.
	MaxClockTimestamp = ClockTimestamp{WallTime: math.MaxInt64, Logical: math.MaxInt32}
	// MinClockTimestamp is the min value allowed for ClockTimestamp.
	MinClockTimestamp = ClockTimestamp{WallTime: 0, Logical: 1}
)

Timestamp constant values.

View Source
var (
	ErrInvalidLengthTimestamp        = fmt.Errorf("proto: negative length found during unmarshaling")
	ErrIntOverflowTimestamp          = fmt.Errorf("proto: integer overflow")
	ErrUnexpectedEndOfGroupTimestamp = fmt.Errorf("proto: unexpected end of group")
)

Functions

func IsUntrustworthyRemoteWallTimeError

func IsUntrustworthyRemoteWallTimeError(err error) bool

IsUntrustworthyRemoteWallTimeError returns true if the error came resulted from a call to Clock.UpdateAndCheckMaxOffset due to the passed ClockTimestamp being too far in the future.

func UnixNano

func UnixNano() int64

UnixNano returns the local machine's physical nanosecond unix epoch timestamp as a convenience to create a HLC via c := hlc.NewClock(hlc.UnixNano, ...).

Types

type Clock

type Clock struct {
	// contains filtered or unexported fields
}

Clock is a hybrid logical clock. Objects of this type model causality while maintaining a relation to physical time. Roughly speaking, timestamps consist of the largest wall clock time among all events, and a logical clock that ticks whenever an event happens in the future of the local physical clock. The data structure is thread safe and thus can safely be shared by multiple goroutines.

See NewClock for details.

func NewClock

func NewClock(physicalClock func() int64, maxOffset time.Duration) *Clock

NewClock creates a new hybrid logical clock associated with the given physical clock. The logical ts is initialized to zero.

The physical clock is typically given by the wall time of the local machine in unix epoch nanoseconds, using hlc.UnixNano. This is not a requirement.

A value of 0 for maxOffset means that clock skew checking, if performed on this clock by RemoteClockMonitor, is disabled.

func (*Clock) MaxOffset

func (c *Clock) MaxOffset() time.Duration

MaxOffset returns the maximal clock offset to any node in the cluster.

A value of 0 means offset checking is disabled.

func (*Clock) Now

func (c *Clock) Now() Timestamp

Now returns a timestamp associated with an event from the local machine that may be sent to other members of the distributed network.

func (*Clock) NowAsClockTimestamp

func (c *Clock) NowAsClockTimestamp() ClockTimestamp

NowAsClockTimestamp is like Now, but returns a ClockTimestamp instead of a raw Timestamp.

This is the counterpart of Update, which is passed a ClockTimestamp received from another member of the distributed network. As such, callers that intend to use the returned timestamp to update a peer's HLC clock should use this method.

func (*Clock) PhysicalNow

func (c *Clock) PhysicalNow() int64

PhysicalNow returns the local wall time.

Note that, contrary to Now(), PhysicalNow does not take into consideration higher clock signals received through Update(). If you want to take them into consideration, use c.Now().GoTime().

func (*Clock) PhysicalTime

func (c *Clock) PhysicalTime() time.Time

PhysicalTime returns a time.Time struct using the local wall time.

func (*Clock) RefreshHLCUpperBound

func (c *Clock) RefreshHLCUpperBound(persistFn func(int64) error, delta int64) error

RefreshHLCUpperBound persists the HLC upper bound and updates the in memory value if the persist succeeds. delta is used to compute the upper bound.

func (*Clock) ResetHLCUpperBound

func (c *Clock) ResetHLCUpperBound(persistFn func(int64) error) error

ResetHLCUpperBound persists a value of 0 as the HLC upper bound which disables upper bound validation

func (*Clock) SleepUntil

func (c *Clock) SleepUntil(ctx context.Context, t Timestamp) error

SleepUntil sleeps until the HLC reaches or exceeds the given timestamp. This typically results in sleeping for the duration between the given timestamp's nanosecond WallTime and the Clock's current WallTime time, but may result in sleeping for longer or shorter, depending on the HLC clock's relation to its physical time source (it may lead it) and whether it advances more rapidly due to updates from other nodes.

If the provided context is canceled, the method will return the cancellation error immediately. If an error is returned, no guarantee is made that the HLC will have reached the specified timestamp.

func (*Clock) StartMonitoringForwardClockJumps

func (c *Clock) StartMonitoringForwardClockJumps(
	ctx context.Context,
	forwardClockJumpCheckEnabledCh <-chan bool,
	tickerFn func(d time.Duration) *time.Ticker,
	tickCallback func(),
) error

StartMonitoringForwardClockJumps starts a goroutine to update the clock's forwardClockJumpCheckEnabled based on the values pushed in forwardClockJumpCheckEnabledCh.

This also keeps lastPhysicalTime up to date to avoid spurious jump errors.

A nil channel or a value of false pushed in forwardClockJumpCheckEnabledCh disables checking clock jumps between two successive reads of the physical clock.

This should only be called once per clock, and will return an error if called more than once

tickerFn is used to create a new ticker

tickCallback is called whenever maxForwardClockJumpCh or a ticker tick is processed

func (*Clock) Update

func (c *Clock) Update(rt ClockTimestamp)

Update takes a hybrid timestamp, usually originating from an event received from another member of a distributed system. The clock is updated to reflect the later of the two. The update does not check the maximum clock offset. To receive an error response instead of forcing the update in case the remote timestamp is too far into the future, use UpdateAndCheckMaxOffset() instead.

func (*Clock) UpdateAndCheckMaxOffset

func (c *Clock) UpdateAndCheckMaxOffset(ctx context.Context, rt ClockTimestamp) error

UpdateAndCheckMaxOffset is like Update, but also takes the wall time into account and returns an error in the event that the supplied remote timestamp exceeds the wall clock time by more than the maximum clock offset.

If an error is returned, it will be detectable with IsUntrustworthyRemoteWallTimeError.

func (*Clock) WallTimeUpperBound

func (c *Clock) WallTimeUpperBound() int64

WallTimeUpperBound returns the in memory value of upper bound to wall time

type ClockSource

type ClockSource struct {
	// contains filtered or unexported fields
}

ClockSource contains the handle of the clock device as well as the clock id.

func MakeClockSource

func MakeClockSource(ctx context.Context, clockDevicePath string) (ClockSource, error)

MakeClockSource creates a new ClockSource for the given device path.

func (ClockSource) UnixNano

func (p ClockSource) UnixNano() int64

UnixNano returns the clock device's physical nanosecond unix epoch timestamp as a convenience to create a HLC via c := hlc.NewClock(dev.UnixNano, ...).

type ClockTimestamp

type ClockTimestamp Timestamp

ClockTimestamp is a Timestamp with the added capability of being able to update a peer's HLC clock. It possesses this capability because the clock timestamp itself is guaranteed to have come from an HLC clock somewhere in the system. As such, a clock timestamp is an promise that some node in the system has a clock with a reading equal to or above its value.

ClockTimestamp is the statically typed version of a Timestamp with its Synthetic flag set to false.

func NewPopulatedClockTimestamp

func NewPopulatedClockTimestamp(r randyTimestamp, easy bool) *ClockTimestamp

NewPopulatedClockTimestamp is needed for the gogoproto.populate option.

func (*ClockTimestamp) Backward

func (t *ClockTimestamp) Backward(s ClockTimestamp)

Backward is like Timestamp.Backward, but for ClockTimestamps.

func (*ClockTimestamp) BackwardWithTimestamp

func (t *ClockTimestamp) BackwardWithTimestamp(s Timestamp)

BackwardWithTimestamp is like Backward, but with a Timestamp parameter.

func (*ClockTimestamp) Equal

func (t *ClockTimestamp) Equal(that interface{}) bool

Equal is needed for the gogoproto.equal option.

func (*ClockTimestamp) Forward

func (t *ClockTimestamp) Forward(s ClockTimestamp) bool

Forward is like Timestamp.Forward, but for ClockTimestamps.

func (ClockTimestamp) IsEmpty

func (t ClockTimestamp) IsEmpty() bool

IsEmpty retruns true if t is an empty ClockTimestamp.

func (ClockTimestamp) Less

Less returns whether the receiver is less than the parameter.

func (ClockTimestamp) LessEq

func (t ClockTimestamp) LessEq(s ClockTimestamp) bool

LessEq returns whether the receiver is less than or equal to the parameter.

func (*ClockTimestamp) MarshalTo

func (t *ClockTimestamp) MarshalTo(data []byte) (int, error)

MarshalTo implements the protoutil.Message interface.

func (*ClockTimestamp) MarshalToSizedBuffer

func (t *ClockTimestamp) MarshalToSizedBuffer(data []byte) (int, error)

MarshalToSizedBuffer implements the protoutil.Message interface.

func (*ClockTimestamp) ProtoMessage

func (t *ClockTimestamp) ProtoMessage()

ProtoMessage implements the protoutil.Message interface.

func (*ClockTimestamp) Reset

func (t *ClockTimestamp) Reset()

Reset implements the protoutil.Message interface.

func (ClockTimestamp) SafeValue

func (t ClockTimestamp) SafeValue()

SafeValue implements the redact.SafeValue interface.

func (*ClockTimestamp) Size

func (t *ClockTimestamp) Size() int

Size implements the protoutil.Message interface.

func (ClockTimestamp) String

func (t ClockTimestamp) String() string

String implements the fmt.Stringer interface.

func (ClockTimestamp) ToTimestamp

func (t ClockTimestamp) ToTimestamp() Timestamp

ToTimestamp upcasts a ClockTimestamp into a Timestamp.

func (*ClockTimestamp) Unmarshal

func (t *ClockTimestamp) Unmarshal(data []byte) error

Unmarshal implements the protoutil.Message interface.

type HybridManualClock

type HybridManualClock struct {
	// contains filtered or unexported fields
}

HybridManualClock is a convenience type to facilitate creating a hybrid logical clock whose physical clock ticks with the wall clock, but that can be moved arbitrarily into the future or paused. HybridManualClock is thread safe.

func NewHybridManualClock

func NewHybridManualClock() *HybridManualClock

NewHybridManualClock returns a new instance, initialized with specified timestamp.

func (*HybridManualClock) Forward

func (m *HybridManualClock) Forward(tsNanos int64)

Forward sets the wall time to the supplied timestamp if this moves the clock forward in time. Note that this takes an absolute timestamp (i.e. a wall clock timestamp), not a delta.

func (*HybridManualClock) Increment

func (m *HybridManualClock) Increment(nanos int64)

Increment increments the hybrid manual clock's timestamp.

func (*HybridManualClock) Pause

func (m *HybridManualClock) Pause()

Pause pauses the hybrid manual clock; the passage of time no longer causes the clock to tick. Increment can still be used, though.

func (*HybridManualClock) Resume

func (m *HybridManualClock) Resume()

Resume resumes the hybrid manual clock; the passage of time continues to cause clock to tick.

func (*HybridManualClock) UnixNano

func (m *HybridManualClock) UnixNano() int64

UnixNano returns the underlying hybrid manual clock's timestamp.

type LegacyTimestamp

type LegacyTimestamp struct {
	// Holds a wall time, typically a unix epoch time expressed in
	// nanoseconds.
	WallTime int64 `protobuf:"varint,1,opt,name=wall_time,json=wallTime" json:"wall_time"`
	// The logical component captures causality for events whose wall
	// times are equal. It is effectively bounded by (maximum clock
	// skew)/(minimal ns between events) and nearly impossible to
	// overflow.
	Logical int32 `protobuf:"varint,2,opt,name=logical" json:"logical"`
	// Indicates that the Timestamp did not come from an HLC clock somewhere
	// in the system and, therefore, does not have the ability to update a
	// peer's HLC clock. If set to true, the "synthetic timestamp" may be
	// arbitrarily disconnected from real time.
	//
	// See the commentary on Timestamp.synthetic for more information.
	//
	// The field is nullable so that it is not serialized when set to false.
	// This ensures that the timestamp encoding does not change across nodes
	// that are and are not aware of this field.
	Synthetic *bool `protobuf:"varint,3,opt,name=synthetic" json:"synthetic,omitempty"`
}

LegacyTimestamp is convertible to hlc.Timestamp, but uses the legacy encoding as it is encoded "below raft".

func NewPopulatedLegacyTimestamp

func NewPopulatedLegacyTimestamp(r randyLegacyTimestamp, easy bool) *LegacyTimestamp

func (*LegacyTimestamp) Descriptor

func (*LegacyTimestamp) Descriptor() ([]byte, []int)

func (LegacyTimestamp) EqOrdering

func (t LegacyTimestamp) EqOrdering(s LegacyTimestamp) bool

EqOrdering returns whether the receiver sorts equally to the parameter.

func (*LegacyTimestamp) Equal

func (this *LegacyTimestamp) Equal(that interface{}) bool

func (LegacyTimestamp) Less

Less returns whether the receiver is less than the parameter.

func (*LegacyTimestamp) Marshal

func (m *LegacyTimestamp) Marshal() (dAtA []byte, err error)

func (*LegacyTimestamp) MarshalTo

func (m *LegacyTimestamp) MarshalTo(dAtA []byte) (int, error)

func (*LegacyTimestamp) MarshalToSizedBuffer

func (m *LegacyTimestamp) MarshalToSizedBuffer(dAtA []byte) (int, error)

func (*LegacyTimestamp) ProtoMessage

func (*LegacyTimestamp) ProtoMessage()

func (*LegacyTimestamp) Reset

func (m *LegacyTimestamp) Reset()

func (LegacyTimestamp) SafeValue

func (LegacyTimestamp) SafeValue()

SafeValue implements the redact.SafeValue interface.

func (*LegacyTimestamp) Size

func (m *LegacyTimestamp) Size() (n int)

func (LegacyTimestamp) String

func (t LegacyTimestamp) String() string

String implements the fmt.Stringer interface.

func (LegacyTimestamp) ToTimestamp

func (t LegacyTimestamp) ToTimestamp() Timestamp

ToTimestamp converts a LegacyTimestamp to a Timestamp.

func (*LegacyTimestamp) Unmarshal

func (m *LegacyTimestamp) Unmarshal(dAtA []byte) error

func (*LegacyTimestamp) XXX_DiscardUnknown

func (m *LegacyTimestamp) XXX_DiscardUnknown()

func (*LegacyTimestamp) XXX_Marshal

func (m *LegacyTimestamp) XXX_Marshal(b []byte, deterministic bool) ([]byte, error)

func (*LegacyTimestamp) XXX_Merge

func (m *LegacyTimestamp) XXX_Merge(src proto.Message)

func (*LegacyTimestamp) XXX_Size

func (m *LegacyTimestamp) XXX_Size() int

func (*LegacyTimestamp) XXX_Unmarshal

func (m *LegacyTimestamp) XXX_Unmarshal(b []byte) error

type ManualClock

type ManualClock struct {
	// contains filtered or unexported fields
}

ManualClock is a convenience type to facilitate creating a hybrid logical clock whose physical clock is manually controlled. ManualClock is thread safe.

func NewManualClock

func NewManualClock(nanos int64) *ManualClock

NewManualClock returns a new instance, initialized with specified timestamp.

func (*ManualClock) Increment

func (m *ManualClock) Increment(incr int64)

Increment atomically increments the manual clock's timestamp.

func (*ManualClock) Set

func (m *ManualClock) Set(nanos int64)

Set atomically sets the manual clock's timestamp.

func (*ManualClock) UnixNano

func (m *ManualClock) UnixNano() int64

UnixNano returns the underlying manual clock's timestamp.

type Timestamp

type Timestamp struct {
	// Holds a wall time, typically a unix epoch time expressed in
	// nanoseconds.
	//
	// It is not safe to mutate this field directly. Instead, use one of the
	// methods on Timestamp, which ensure that the synthetic flag is updated
	// appropriately.
	WallTime int64 `protobuf:"varint,1,opt,name=wall_time,json=wallTime,proto3" json:"wall_time,omitempty"`
	// The logical component captures causality for events whose wall times
	// are equal. It is effectively bounded by (maximum clock skew)/(minimal
	// ns between events) and nearly impossible to overflow.
	//
	// It is not safe to mutate this field directly. Instead, use one of the
	// methods on Timestamp, which ensure that the synthetic flag is updated
	// appropriately.
	Logical int32 `protobuf:"varint,2,opt,name=logical,proto3" json:"logical,omitempty"`
	// Indicates that the Timestamp did not come from an HLC clock somewhere
	// in the system and, therefore, does not have the ability to update a
	// peer's HLC clock. If set to true, the "synthetic timestamp" may be
	// arbitrarily disconnected from real time.
	//
	// The flag serves as the dynamically typed version of a ClockTimestamp
	// (but inverted). Only Timestamps with this flag set to false can be
	// downcast to a ClockTimestamp successfully (see TryToClockTimestamp).
	//
	// Synthetic timestamps with this flag set to true are central to
	// non-blocking transactions, which write "into the future". Setting the
	// flag to true is also used to disconnect some committed MVCC versions
	// from observed timestamps by indicating that those versions were moved
	// from the timestamp at which they were originally written. Committed
	// MVCC versions with synthetic timestamps require observing the full
	// uncertainty interval, whereas readings off the leaseholders's clock
	// can tighten the uncertainty interval that is applied to MVCC versions
	// with clock timestamp.
	//
	// This flag does not affect the sort order of Timestamps. However, it
	// is considered when performing structural equality checks (e.g. using
	// the == operator). Consider use of the EqOrdering method when testing
	// for equality.
	Synthetic bool `protobuf:"varint,3,opt,name=synthetic,proto3" json:"synthetic,omitempty"`
}

Timestamp represents a state of the hybrid logical clock.

func NewPopulatedTimestamp

func NewPopulatedTimestamp(r randyTimestamp, easy bool) *Timestamp

func ParseTimestamp

func ParseTimestamp(str string) (_ Timestamp, err error)

ParseTimestamp attempts to parse the string generated from Timestamp.String().

func (Timestamp) Add

func (t Timestamp) Add(wallTime int64, logical int32) Timestamp

Add returns a timestamp with the WallTime and Logical components increased. wallTime is expressed in nanos.

TODO(nvanbenschoten): consider an AddNanos method that takes a time.Duration.

func (Timestamp) AsOfSystemTime

func (t Timestamp) AsOfSystemTime() string

AsOfSystemTime returns a string to be used in an AS OF SYSTEM TIME query.

func (*Timestamp) Backward

func (t *Timestamp) Backward(s Timestamp)

Backward replaces the receiver with the argument, if that moves it backwards in time.

func (Timestamp) Clone

func (t Timestamp) Clone() *Timestamp

Clone return a new timestamp that has the same contents as the receiver.

func (Timestamp) Compare

func (t Timestamp) Compare(s Timestamp) int

Compare returns -1 if this timestamp is lesser than the given timestamp, 1 if it is greater, and 0 if they are equal.

func (*Timestamp) Descriptor

func (*Timestamp) Descriptor() ([]byte, []int)

func (Timestamp) EqOrdering

func (t Timestamp) EqOrdering(s Timestamp) bool

EqOrdering returns whether the receiver sorts equally to the parameter.

This method is split from tests of structural equality (Equal and the equals operator) because it does not consider differences in flags and only considers whether the walltime and logical time differ between the timestamps.

func (*Timestamp) Equal

func (this *Timestamp) Equal(that interface{}) bool

func (Timestamp) FloorPrev

func (t Timestamp) FloorPrev() Timestamp

FloorPrev returns a timestamp earlier than the current timestamp. If it can subtract a logical tick without wrapping around, it does so. Otherwise it subtracts a nanosecond from the walltime.

func (*Timestamp) Forward

func (t *Timestamp) Forward(s Timestamp) bool

Forward replaces the receiver with the argument, if that moves it forwards in time. Returns true if the timestamp was adjusted to a larger time and false otherwise.

func (Timestamp) GoTime

func (t Timestamp) GoTime() time.Time

GoTime converts the timestamp to a time.Time.

func (Timestamp) IsEmpty

func (t Timestamp) IsEmpty() bool

IsEmpty returns true if t is an empty Timestamp.

func (Timestamp) IsSet

func (t Timestamp) IsSet() bool

IsSet returns true if t is not an empty Timestamp.

func (Timestamp) Less

func (t Timestamp) Less(s Timestamp) bool

Less returns whether the receiver is less than the parameter.

func (Timestamp) LessEq

func (t Timestamp) LessEq(s Timestamp) bool

LessEq returns whether the receiver is less than or equal to the parameter.

func (*Timestamp) Marshal

func (m *Timestamp) Marshal() (dAtA []byte, err error)

func (*Timestamp) MarshalTo

func (m *Timestamp) MarshalTo(dAtA []byte) (int, error)

func (*Timestamp) MarshalToSizedBuffer

func (m *Timestamp) MarshalToSizedBuffer(dAtA []byte) (int, error)

func (Timestamp) Next

func (t Timestamp) Next() Timestamp

Next returns the timestamp with the next later timestamp.

func (Timestamp) Prev

func (t Timestamp) Prev() Timestamp

Prev returns the next earliest timestamp.

func (*Timestamp) ProtoMessage

func (*Timestamp) ProtoMessage()

func (*Timestamp) Reset

func (m *Timestamp) Reset()

func (Timestamp) SafeValue

func (Timestamp) SafeValue()

SafeValue implements the redact.SafeValue interface.

func (*Timestamp) Size

func (m *Timestamp) Size() (n int)

func (Timestamp) String

func (t Timestamp) String() string

String implements the fmt.Stringer interface.

func (Timestamp) ToLegacyTimestamp

func (t Timestamp) ToLegacyTimestamp() LegacyTimestamp

ToLegacyTimestamp converts a Timestamp to a LegacyTimestamp.

func (Timestamp) TryToClockTimestamp

func (t Timestamp) TryToClockTimestamp() (ClockTimestamp, bool)

TryToClockTimestamp attempts to downcast a Timestamp into a ClockTimestamp. Returns the result and a boolean indicating whether the cast succeeded.

func (*Timestamp) Unmarshal

func (m *Timestamp) Unmarshal(dAtA []byte) error

func (Timestamp) UnsafeToClockTimestamp

func (t Timestamp) UnsafeToClockTimestamp() ClockTimestamp

UnsafeToClockTimestamp converts a Timestamp to a ClockTimestamp, regardless of whether such a cast would be legal according to the Synthetic flag. The method should only be used in tests.

func (Timestamp) WallPrev

func (t Timestamp) WallPrev() Timestamp

WallPrev subtracts 1 from the WallTime and resets Logical.

func (Timestamp) WithSynthetic

func (t Timestamp) WithSynthetic(val bool) Timestamp

WithSynthetic returns a timestamp with the Synthetic flag set to val.

func (*Timestamp) XXX_DiscardUnknown

func (m *Timestamp) XXX_DiscardUnknown()

func (*Timestamp) XXX_Marshal

func (m *Timestamp) XXX_Marshal(b []byte, deterministic bool) ([]byte, error)

func (*Timestamp) XXX_Merge

func (m *Timestamp) XXX_Merge(src proto.Message)

func (*Timestamp) XXX_Size

func (m *Timestamp) XXX_Size() int

func (*Timestamp) XXX_Unmarshal

func (m *Timestamp) XXX_Unmarshal(b []byte) error

Jump to

Keyboard shortcuts

? : This menu
/ : Search site
f or F : Jump to
y or Y : Canonical URL