protocol

package
v0.89.0 Latest Latest
Warning

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

Go to latest
Published: Sep 2, 2021 License: MIT Imports: 33 Imported by: 21

Documentation

Overview

Package protocol defines the core broker datamodel, validation behaviors, and gRPC APIs which are shared across clients and broker servers. Datamodel types and APIs are implemented as generated protobuf messages and stubs, typically extended with additional parsing, validation, and shared implementation behaviors. A central goal of this package to be highly exacting in defining allowed "shapes" that types & messages may take (through implementations of the Validator interface), providing strong guarantees to brokers and clients that messages are well-formed without need for additional ad-hoc, repetitive checks (which often become a maintenance burden).

The package also implements a gRPC "dispatcher" load balancer which provides a tight integration between available routes of a journal and gRPC's selection of an appropriate transport for a given RPC. Use of the balancer, identified by DispatcherGRPCBalancerName, allows clients and servers to use a single *grpc.ClientConn through which all RPCs are dispatched. Context metadata, attached via WithDispatch*(), informs the balancer of the set of specific servers suitable for serving a request. The balancer can factor considerations such as which servers have ready transports, or whether the RPC will cross availability zones to make a final transport selection.

By convention, this package is usually imported as `pb`, short for "Protocol of Broker", due to it's ubiquity and to distinguish it from package go.gazette.dev/core/consumer/protocol (imported as `pc`). Eg,

import pb "go.gazette.dev/core/broker/protocol"

Index

Constants

View Source
const DispatcherGRPCBalancerName = "protocolDispatcher"

DispatcherGRPCBalancerName is the client-side dispatcher's registered gRPC balancer. To utilize client-side dispatching, the service endpoint should be dialed with grpc.WithBalancerName(protocol.DispatcherGRPCBalancerName).

View Source
const (
	// TokenSymbols is allowed runes of "tokens",
	// Note that any character with ordinal value less than or equal to '#' (35),
	// which is the allocator KeySpace separator, must not be included in this alphabet.
	// The alphabet leads with '-' to facilitate escaping in |reToken|.
	TokenSymbols = "-_+/."
)

Variables

View Source
var (
	ErrInvalidLengthProtocol        = fmt.Errorf("proto: negative length found during unmarshaling")
	ErrIntOverflowProtocol          = fmt.Errorf("proto: integer overflow")
	ErrUnexpectedEndOfGroupProtocol = fmt.Errorf("proto: unexpected end of group")
)
View Source
var CompressionCodec_name = map[int32]string{
	0: "INVALID",
	1: "NONE",
	2: "GZIP",
	3: "ZSTANDARD",
	4: "SNAPPY",
	5: "GZIP_OFFLOAD_DECOMPRESSION",
}
View Source
var CompressionCodec_value = map[string]int32{
	"INVALID":                    0,
	"NONE":                       1,
	"GZIP":                       2,
	"ZSTANDARD":                  3,
	"SNAPPY":                     4,
	"GZIP_OFFLOAD_DECOMPRESSION": 5,
}
View Source
var JournalSpec_Flag_name = map[int32]string{
	0: "NOT_SPECIFIED",
	1: "O_RDONLY",
	2: "O_WRONLY",
	4: "O_RDWR",
}
View Source
var JournalSpec_Flag_value = map[string]int32{
	"NOT_SPECIFIED": 0,
	"O_RDONLY":      1,
	"O_WRONLY":      2,
	"O_RDWR":        4,
}
View Source
var Status_name = map[int32]string{
	0:  "OK",
	1:  "JOURNAL_NOT_FOUND",
	2:  "NO_JOURNAL_PRIMARY_BROKER",
	3:  "NOT_JOURNAL_PRIMARY_BROKER",
	5:  "NOT_JOURNAL_BROKER",
	4:  "INSUFFICIENT_JOURNAL_BROKERS",
	6:  "OFFSET_NOT_YET_AVAILABLE",
	7:  "WRONG_ROUTE",
	8:  "PROPOSAL_MISMATCH",
	9:  "ETCD_TRANSACTION_FAILED",
	10: "NOT_ALLOWED",
	11: "WRONG_APPEND_OFFSET",
	12: "INDEX_HAS_GREATER_OFFSET",
	13: "REGISTER_MISMATCH",
}
View Source
var Status_value = map[string]int32{
	"OK":                           0,
	"JOURNAL_NOT_FOUND":            1,
	"NO_JOURNAL_PRIMARY_BROKER":    2,
	"NOT_JOURNAL_PRIMARY_BROKER":   3,
	"NOT_JOURNAL_BROKER":           5,
	"INSUFFICIENT_JOURNAL_BROKERS": 4,
	"OFFSET_NOT_YET_AVAILABLE":     6,
	"WRONG_ROUTE":                  7,
	"PROPOSAL_MISMATCH":            8,
	"ETCD_TRANSACTION_FAILED":      9,
	"NOT_ALLOWED":                  10,
	"WRONG_APPEND_OFFSET":          11,
	"INDEX_HAS_GREATER_OFFSET":     12,
	"REGISTER_MISMATCH":            13,
}

Functions

func ExtendContext

func ExtendContext(err error, format string, args ...interface{}) error

ExtendContext type-checks |err| to a *ValidationError, and if matched extends it with |context|. In all cases the value of |err| is returned.

func NewValidationError

func NewValidationError(format string, args ...interface{}) error

NewValidationError parallels fmt.Errorf to returns a new ValidationError instance.

func RegisterGRPCDispatcher

func RegisterGRPCDispatcher(localZone string)

RegisterGRPCDispatcher registers the dispatcher balancer with gRPC. It should be called once at program startup. The supplied |localZone| is used to prefer intra-zone (over inter-zone) members where able.

func RegisterJournalServer

func RegisterJournalServer(s *grpc.Server, srv JournalServer)

func ValidatePathComponent added in v0.85.1

func ValidatePathComponent(n string, min, max int) error

ValidatePathComponent ensures the string is of length [min, max], that it is a "clean" path (as defined by path.Clean), is non-rooted, and consists only of characters drawn from pathSymbols.

func ValidateSingleValueLabels

func ValidateSingleValueLabels(m LabelSet) error

ValidateSingleValueLabels compares the LabelSet to labels.SingleValueLabels, and returns an error if any labels have multiple values.

func ValidateToken

func ValidateToken(n, symbols string, min, max int) error

ValidateToken ensures the string is of length [min, max] and consists only of runes drawn from a restricted set: unicode.Letter and unicode.Digit character classes, and passed allowed symbols. Tokens are simple strings which represent things like member zones, suffixes, label names, and values. They generally do not allow spaces.

func WithDispatchDefault

func WithDispatchDefault(ctx context.Context) context.Context

WithDispatchDefault attaches a Route and ProcessSpec_ID which indicate that the RPC should be dispatched to the default service address.

func WithDispatchItemRoute

func WithDispatchItemRoute(ctx context.Context, dr DispatchRouter, item string, requirePrimary bool) context.Context

WithDispatchItemRoute uses the DispatchRouter to resolve |item| to a Route and ProcessSpec_ID, which are in-turn attached to the Context and returned for dispatcher's use.

func WithDispatchRoute

func WithDispatchRoute(ctx context.Context, rt Route, id ProcessSpec_ID) context.Context

WithDispatchRoute attaches a Route and optional ProcessSpec_ID to a Context passed to a gRPC RPC call. If ProcessSpec_ID is non-zero valued, the RPC is dispatched to the specified member. Otherwise, the RPC is dispatched to a Route member, preferring:

  • A member not having a currently-broken network connection (eg, due to a stale Route or network split).
  • A member which is in the same zone as the caller (potentially reducing network traffic costs.
  • A member having a Ready connection (potentially reducing latency).

Types

type AppendRequest

type AppendRequest struct {
	// Header is attached by a proxying broker peer to the first AppendRequest
	// message.
	Header *Header `protobuf:"bytes,1,opt,name=header,proto3" json:"header,omitempty"`
	// Journal to be appended to.
	Journal Journal `protobuf:"bytes,2,opt,name=journal,proto3,casttype=Journal" json:"journal,omitempty"`
	// If do_not_proxy is true, the broker will not proxy the append if it is
	// not the current primary.
	DoNotProxy bool `protobuf:"varint,3,opt,name=do_not_proxy,json=doNotProxy,proto3" json:"do_not_proxy,omitempty"`
	// Journal offset at which the append should begin. Most clients should leave
	// at zero, which uses the broker's tracked offset. The append offset must be
	// one greater than furthest written offset of the journal, or
	// WRONG_APPEND_OFFSET is returned.
	Offset Offset `protobuf:"varint,5,opt,name=offset,proto3,casttype=Offset" json:"offset,omitempty"`
	// Selector of journal registers which must be satisfied for the request
	// to proceed. If not matched, the RPC is failed with REGISTER_MISMATCH.
	//
	// There's one important exception: if the set of registers associated with
	// a journal is completely empty, then *any* selector is considered as
	// matching. While perhaps surprising, this behavior supports the intended
	// use of registers for cooperative locking, whereby an empty set of
	// registers can be thought of as an "unlocked" state. More practically, if
	// Etcd consensus is lost then so are current register values: on recovery
	// journals will restart with an empty set. This behavior ensures that an
	// existing process holding a prior lock can continue to write -- at least
	// until another process updates registers once again.
	CheckRegisters *LabelSelector `protobuf:"bytes,6,opt,name=check_registers,json=checkRegisters,proto3" json:"check_registers,omitempty"`
	// Labels to union with current registers if the RPC succeeds and appended
	// at least one byte.
	UnionRegisters *LabelSet `protobuf:"bytes,7,opt,name=union_registers,json=unionRegisters,proto3" json:"union_registers,omitempty"`
	// Labels to subtract from current registers if the RPC succeeds and appended
	// at least one byte.
	SubtractRegisters *LabelSet `protobuf:"bytes,8,opt,name=subtract_registers,json=subtractRegisters,proto3" json:"subtract_registers,omitempty"`
	// Content chunks to be appended. Immediately prior to closing the stream,
	// the client must send an empty chunk (eg, zero-valued AppendRequest) to
	// indicate the Append should be committed. Absence of this empty chunk
	// prior to EOF is interpreted by the broker as a rollback of the Append.
	Content []byte `protobuf:"bytes,4,opt,name=content,proto3" json:"content,omitempty"`
}

AppendRequest is the streamed request message of the broker Append RPC. Append request streams consist of an initial message having all parameters of the append, such as the journal to append to and preconditions, followed by an unbounded number of messages having only content "chunks".

It's not required that the appender know the append size when starting the Append RPC -- rather, the client indicates the stream is complete by sending a final, empty "chunk" message. However be aware that the broker will aggressively time out stalled Append clients, and clients should not start RPCs until all content chunks are available for immediate writing.

Append RPCs also expose a concept of journal "registers": LabelSets which participate in the journal's transactional append machinery. Note that registers are sent and verified with every replicated journal transaction, so they're _really_ intended to be very small.

Append RPCs may upsert (union) or delete (subtract) labels from the journal's registers. Register consensus is achieved by piggy-backing on the append itself: if peers disagree, the registers of the replica having the largest journal byte offset always win. For this reason, only RPCs appending at least one byte may modify registers.

Append RPCs may also require that registers match an arbitrary selector before the RPC may proceed. For example, a write fence can be implemented by requiring that a "author" register is of an expected value. At-most-once semantics can be implemented as a check-and-set over a single register.

Also be aware that a register update can still occur even for RPCs which are reported as failed to the client. That's because an append RPC succeeds only after all replicas acknowledge it, but a RPC which applies to some replicas but not all still moves the journal offset forward, and therefore updates journal registers.

func (*AppendRequest) Descriptor

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

func (*AppendRequest) Equal added in v0.83.1

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

func (*AppendRequest) Marshal

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

func (*AppendRequest) MarshalTo

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

func (*AppendRequest) MarshalToSizedBuffer added in v0.86.1

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

func (*AppendRequest) ProtoMessage

func (*AppendRequest) ProtoMessage()

func (*AppendRequest) ProtoSize

func (m *AppendRequest) ProtoSize() (n int)

func (*AppendRequest) Reset

func (m *AppendRequest) Reset()

func (*AppendRequest) String

func (m *AppendRequest) String() string

func (*AppendRequest) Unmarshal

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

func (*AppendRequest) Validate

func (m *AppendRequest) Validate() error

Validate returns an error if the AppendRequest is not well-formed.

func (*AppendRequest) XXX_DiscardUnknown

func (m *AppendRequest) XXX_DiscardUnknown()

func (*AppendRequest) XXX_Marshal

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

func (*AppendRequest) XXX_Merge

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

func (*AppendRequest) XXX_Size

func (m *AppendRequest) XXX_Size() int

func (*AppendRequest) XXX_Unmarshal

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

type AppendResponse

type AppendResponse struct {
	// Status of the Append RPC.
	Status Status `protobuf:"varint,1,opt,name=status,proto3,enum=protocol.Status" json:"status,omitempty"`
	// Header of the response.
	Header Header `protobuf:"bytes,2,opt,name=header,proto3" json:"header"`
	// If status is OK, then |commit| is the Fragment which places the
	// committed Append content within the Journal.
	Commit *Fragment `protobuf:"bytes,3,opt,name=commit,proto3" json:"commit,omitempty"`
	// Current registers of the journal.
	Registers *LabelSet `protobuf:"bytes,4,opt,name=registers,proto3" json:"registers,omitempty"`
	// Total number of RPC content chunks processed in this append.
	TotalChunks int64 `protobuf:"varint,5,opt,name=total_chunks,json=totalChunks,proto3" json:"total_chunks,omitempty"`
	// Number of content chunks which were delayed by journal flow control.
	DelayedChunks int64 `protobuf:"varint,6,opt,name=delayed_chunks,json=delayedChunks,proto3" json:"delayed_chunks,omitempty"`
}

AppendResponse is the unary response message of the broker Append RPC.

func (*AppendResponse) Descriptor

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

func (*AppendResponse) Equal added in v0.83.1

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

func (*AppendResponse) Marshal

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

func (*AppendResponse) MarshalTo

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

func (*AppendResponse) MarshalToSizedBuffer added in v0.86.1

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

func (*AppendResponse) ProtoMessage

func (*AppendResponse) ProtoMessage()

func (*AppendResponse) ProtoSize

func (m *AppendResponse) ProtoSize() (n int)

func (*AppendResponse) Reset

func (m *AppendResponse) Reset()

func (*AppendResponse) String

func (m *AppendResponse) String() string

func (*AppendResponse) Unmarshal

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

func (*AppendResponse) Validate

func (m *AppendResponse) Validate() error

Validate returns an error if the AppendResponse is not well-formed.

func (*AppendResponse) XXX_DiscardUnknown

func (m *AppendResponse) XXX_DiscardUnknown()

func (*AppendResponse) XXX_Marshal

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

func (*AppendResponse) XXX_Merge

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

func (*AppendResponse) XXX_Size

func (m *AppendResponse) XXX_Size() int

func (*AppendResponse) XXX_Unmarshal

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

type ApplyRequest

type ApplyRequest struct {
	Changes []ApplyRequest_Change `protobuf:"bytes,1,rep,name=changes,proto3" json:"changes"`
}

ApplyRequest is the unary request message of the broker Apply RPC.

func (*ApplyRequest) Descriptor

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

func (*ApplyRequest) Equal added in v0.83.1

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

func (*ApplyRequest) Marshal

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

func (*ApplyRequest) MarshalTo

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

func (*ApplyRequest) MarshalToSizedBuffer added in v0.86.1

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

func (*ApplyRequest) ProtoMessage

func (*ApplyRequest) ProtoMessage()

func (*ApplyRequest) ProtoSize

func (m *ApplyRequest) ProtoSize() (n int)

func (*ApplyRequest) Reset

func (m *ApplyRequest) Reset()

func (*ApplyRequest) String

func (m *ApplyRequest) String() string

func (*ApplyRequest) Unmarshal

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

func (*ApplyRequest) Validate

func (m *ApplyRequest) Validate() error

func (*ApplyRequest) XXX_DiscardUnknown

func (m *ApplyRequest) XXX_DiscardUnknown()

func (*ApplyRequest) XXX_Marshal

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

func (*ApplyRequest) XXX_Merge

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

func (*ApplyRequest) XXX_Size

func (m *ApplyRequest) XXX_Size() int

func (*ApplyRequest) XXX_Unmarshal

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

type ApplyRequest_Change

type ApplyRequest_Change struct {
	// Expected ModRevision of the current JournalSpec. If the Journal is being
	// created, expect_mod_revision is zero.
	ExpectModRevision int64 `protobuf:"varint,1,opt,name=expect_mod_revision,json=expectModRevision,proto3" json:"expect_mod_revision,omitempty"`
	// JournalSpec to be updated (if expect_mod_revision > 0) or created
	// (if expect_mod_revision == 0).
	Upsert *JournalSpec `protobuf:"bytes,2,opt,name=upsert,proto3" json:"upsert,omitempty"`
	// Journal to be deleted. expect_mod_revision must not be zero.
	Delete Journal `protobuf:"bytes,3,opt,name=delete,proto3,casttype=Journal" json:"delete,omitempty"`
}

Change defines an insertion, update, or deletion to be applied to the set of JournalSpecs. Exactly one of |upsert| or |delete| must be set.

func (*ApplyRequest_Change) Descriptor

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

func (*ApplyRequest_Change) Equal added in v0.83.1

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

func (*ApplyRequest_Change) Marshal

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

func (*ApplyRequest_Change) MarshalTo

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

func (*ApplyRequest_Change) MarshalToSizedBuffer added in v0.86.1

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

func (*ApplyRequest_Change) ProtoMessage

func (*ApplyRequest_Change) ProtoMessage()

func (*ApplyRequest_Change) ProtoSize

func (m *ApplyRequest_Change) ProtoSize() (n int)

func (*ApplyRequest_Change) Reset

func (m *ApplyRequest_Change) Reset()

func (*ApplyRequest_Change) String

func (m *ApplyRequest_Change) String() string

func (*ApplyRequest_Change) Unmarshal

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

func (*ApplyRequest_Change) Validate

func (m *ApplyRequest_Change) Validate() error

func (*ApplyRequest_Change) XXX_DiscardUnknown

func (m *ApplyRequest_Change) XXX_DiscardUnknown()

func (*ApplyRequest_Change) XXX_Marshal

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

func (*ApplyRequest_Change) XXX_Merge

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

func (*ApplyRequest_Change) XXX_Size

func (m *ApplyRequest_Change) XXX_Size() int

func (*ApplyRequest_Change) XXX_Unmarshal

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

type ApplyResponse

type ApplyResponse struct {
	// Status of the Apply RPC.
	Status Status `protobuf:"varint,1,opt,name=status,proto3,enum=protocol.Status" json:"status,omitempty"`
	// Header of the response.
	Header Header `protobuf:"bytes,2,opt,name=header,proto3" json:"header"`
}

ApplyResponse is the unary response message of the broker Apply RPC.

func (*ApplyResponse) Descriptor

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

func (*ApplyResponse) Equal added in v0.83.1

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

func (*ApplyResponse) Marshal

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

func (*ApplyResponse) MarshalTo

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

func (*ApplyResponse) MarshalToSizedBuffer added in v0.86.1

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

func (*ApplyResponse) ProtoMessage

func (*ApplyResponse) ProtoMessage()

func (*ApplyResponse) ProtoSize

func (m *ApplyResponse) ProtoSize() (n int)

func (*ApplyResponse) Reset

func (m *ApplyResponse) Reset()

func (*ApplyResponse) String

func (m *ApplyResponse) String() string

func (*ApplyResponse) Unmarshal

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

func (*ApplyResponse) Validate

func (m *ApplyResponse) Validate() error

func (*ApplyResponse) XXX_DiscardUnknown

func (m *ApplyResponse) XXX_DiscardUnknown()

func (*ApplyResponse) XXX_Marshal

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

func (*ApplyResponse) XXX_Merge

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

func (*ApplyResponse) XXX_Size

func (m *ApplyResponse) XXX_Size() int

func (*ApplyResponse) XXX_Unmarshal

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

type BrokerSpec

type BrokerSpec struct {
	// ProcessSpec of the broker.
	ProcessSpec `protobuf:"bytes,1,opt,name=process_spec,json=processSpec,proto3,embedded=process_spec" json:"process_spec" yaml:",inline"`
	// Maximum number of assigned Journal replicas.
	JournalLimit uint32 `protobuf:"varint,2,opt,name=journal_limit,json=journalLimit,proto3" json:"journal_limit,omitempty"`
}

BrokerSpec describes a Gazette broker and its configuration.

func (*BrokerSpec) Descriptor

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

func (*BrokerSpec) Equal added in v0.83.1

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

func (*BrokerSpec) ItemLimit

func (m *BrokerSpec) ItemLimit() int

v3_allocator.MemberValue implementation.

func (*BrokerSpec) Marshal

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

func (*BrokerSpec) MarshalString

func (m *BrokerSpec) MarshalString() string

MarshalString returns the marshaled encoding of the JournalSpec as a string.

func (*BrokerSpec) MarshalTo

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

func (*BrokerSpec) MarshalToSizedBuffer added in v0.86.1

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

func (*BrokerSpec) ProtoMessage

func (*BrokerSpec) ProtoMessage()

func (*BrokerSpec) ProtoSize

func (m *BrokerSpec) ProtoSize() (n int)

func (*BrokerSpec) Reset

func (m *BrokerSpec) Reset()

func (*BrokerSpec) String

func (m *BrokerSpec) String() string

func (*BrokerSpec) Unmarshal

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

func (*BrokerSpec) Validate

func (m *BrokerSpec) Validate() error

Validate returns an error if the BrokerSpec is not well-formed.

func (*BrokerSpec) XXX_DiscardUnknown

func (m *BrokerSpec) XXX_DiscardUnknown()

func (*BrokerSpec) XXX_Marshal

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

func (*BrokerSpec) XXX_Merge

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

func (*BrokerSpec) XXX_Size

func (m *BrokerSpec) XXX_Size() int

func (*BrokerSpec) XXX_Unmarshal

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

func (*BrokerSpec) ZeroLimit

func (m *BrokerSpec) ZeroLimit()

ZeroLimit zeros the BrokerSpec JournalLimit.

type CompressionCodec

type CompressionCodec int32

CompressionCode defines codecs known to Gazette.

const (
	// INVALID is the zero-valued CompressionCodec, and is not a valid codec.
	CompressionCodec_INVALID CompressionCodec = 0
	// NONE encodes Fragments without any applied compression, with default suffix
	// ".raw".
	CompressionCodec_NONE CompressionCodec = 1
	// GZIP encodes Fragments using the Gzip library, with default suffix ".gz".
	CompressionCodec_GZIP CompressionCodec = 2
	// ZSTANDARD encodes Fragments using the ZStandard library, with default
	// suffix ".zst".
	CompressionCodec_ZSTANDARD CompressionCodec = 3
	// SNAPPY encodes Fragments using the Snappy library, with default suffix
	// ".sz".
	CompressionCodec_SNAPPY CompressionCodec = 4
	// GZIP_OFFLOAD_DECOMPRESSION is the GZIP codec with additional behavior
	// around reads and writes to remote Fragment stores, designed to offload
	// the work of decompression onto compatible stores. Specifically:
	//  * Fragments are written with a "Content-Encoding: gzip" header.
	//  * Client read requests are made with "Accept-Encoding: identity".
	// This can be helpful in contexts where reader IO bandwidth to the storage
	// API is unconstrained, as the cost of decompression is offloaded to the
	// store and CPU-intensive batch readers may receive a parallelism benefit.
	// While this codec may provide substantial read-time performance
	// improvements, it is an advanced configuration and the "Content-Encoding"
	// header handling can be subtle and sometimes confusing. It uses the default
	// suffix ".gzod".
	CompressionCodec_GZIP_OFFLOAD_DECOMPRESSION CompressionCodec = 5
)

func CompressionCodecFromExtension

func CompressionCodecFromExtension(ext string) (CompressionCodec, error)

CompressionCodecFromExtension matches a file extension to its corresponding CompressionCodec.

func (CompressionCodec) EnumDescriptor

func (CompressionCodec) EnumDescriptor() ([]byte, []int)

func (CompressionCodec) MarshalYAML

func (m CompressionCodec) MarshalYAML() (interface{}, error)

func (CompressionCodec) String

func (x CompressionCodec) String() string

func (CompressionCodec) ToExtension

func (m CompressionCodec) ToExtension() string

ToExtension returns the file extension of the CompressionCodec.

func (*CompressionCodec) UnmarshalYAML

func (m *CompressionCodec) UnmarshalYAML(unmarshal func(interface{}) error) error

func (CompressionCodec) Validate

func (m CompressionCodec) Validate() error

Validate returns an error if the CompressionCodec is not well-formed.

type DispatchRouter

type DispatchRouter interface {
	// Route an |item| to a Route, which may be empty if the Route is unknown.
	Route(ctx context.Context, item string) Route
	// UpdateRoute for |item|. A nil |route| is treated as an invalidation.
	UpdateRoute(item string, route *Route)
	// IsNoopRouter returns true if Route is a no-op.
	IsNoopRouter() bool
}

DispatchRouter routes item to Routes, and observes item Routes.

type Endpoint

type Endpoint string

Endpoint defines an accessible service address. It is a URL, where the scheme defines the network transport and semantics of the host, path, and query components. At present, supported schemes are:

func (Endpoint) URL

func (ep Endpoint) URL() *url.URL

URL returns the Endpoint as a URL. The Endpoint must Validate, or URL panics.

func (Endpoint) Validate

func (ep Endpoint) Validate() error

Validate returns an error if the Endpoint is not well-formed.

type Fragment

type Fragment struct {
	// Journal of the Fragment.
	Journal Journal `protobuf:"bytes,1,opt,name=journal,proto3,casttype=Journal" json:"journal,omitempty"`
	// Begin (inclusive) and end (exclusive) offset of the Fragment within the
	// Journal.
	Begin Offset `protobuf:"varint,2,opt,name=begin,proto3,casttype=Offset" json:"begin,omitempty"`
	End   Offset `protobuf:"varint,3,opt,name=end,proto3,casttype=Offset" json:"end,omitempty"`
	// SHA1 sum of the Fragment's content.
	Sum SHA1Sum `protobuf:"bytes,4,opt,name=sum,proto3" json:"sum"`
	// Codec with which the Fragment's content is compressed.
	CompressionCodec CompressionCodec `` /* 141-byte string literal not displayed */
	// Fragment store which backs the Fragment. Empty if the Fragment has yet to
	// be persisted and is still local to a Broker.
	BackingStore FragmentStore `protobuf:"bytes,6,opt,name=backing_store,json=backingStore,proto3,casttype=FragmentStore" json:"backing_store,omitempty"`
	// Modification timestamp of the Fragment within the backing store,
	// represented as seconds since the epoch.
	ModTime int64 `protobuf:"varint,7,opt,name=mod_time,json=modTime,proto3" json:"mod_time,omitempty"`
	// Path postfix under which the fragment is persisted to the store.
	// The complete Fragment store path is built from any path components of the
	// backing store, followed by the journal name, followed by the path postfix.
	PathPostfix string `protobuf:"bytes,8,opt,name=path_postfix,json=pathPostfix,proto3" json:"path_postfix,omitempty"`
}

Fragment is a content-addressed description of a contiguous Journal span, defined by the [begin, end) offset range covered by the Fragment and the SHA1 sum of the corresponding Journal content.

func ParseFragmentFromRelativePath added in v0.85.1

func ParseFragmentFromRelativePath(journal Journal, name string) (Fragment, error)

ParseFragmentFromRelativePath parses a Fragment from its relative path name, under the Journal's storage location within a fragment store. Path components contributed by the Journal must have already been stripped from the path string, leaving only a path postfix, content name, and compression extension.

ParseFragmentFromRelativePath("a/journal",
    "a=1/b=2/00000000499602d2-7fffffffffffffff-0102030405060708090a0b0c0d0e0f1011121314.gz")

func (*Fragment) ContentLength

func (m *Fragment) ContentLength() int64

ContentLength returns the number of content bytes contained in this Fragment. If compression is used, this will differ from the file size of the Fragment.

func (*Fragment) ContentName

func (m *Fragment) ContentName() string

ContentName returns the content-addressed base file name of this Fragment.

func (*Fragment) ContentPath

func (m *Fragment) ContentPath() string

ContentPath returns the content-addressed path of this Fragment.

func (*Fragment) Descriptor

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

func (*Fragment) Equal added in v0.83.1

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

func (*Fragment) Marshal

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

func (*Fragment) MarshalTo

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

func (*Fragment) MarshalToSizedBuffer added in v0.86.1

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

func (*Fragment) ProtoMessage

func (*Fragment) ProtoMessage()

func (*Fragment) ProtoSize

func (m *Fragment) ProtoSize() (n int)

func (*Fragment) Reset

func (m *Fragment) Reset()

func (*Fragment) String

func (m *Fragment) String() string

func (*Fragment) Unmarshal

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

func (*Fragment) Validate

func (m *Fragment) Validate() error

Validate returns an error if the Fragment is not well-formed.

func (*Fragment) XXX_DiscardUnknown

func (m *Fragment) XXX_DiscardUnknown()

func (*Fragment) XXX_Marshal

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

func (*Fragment) XXX_Merge

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

func (*Fragment) XXX_Size

func (m *Fragment) XXX_Size() int

func (*Fragment) XXX_Unmarshal

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

type FragmentStore

type FragmentStore string

FragmentStore defines a storage backend base path for Journal Fragments. It is a URL, where the scheme defines the storage backend service. As FragmentStores "root" remote storage locations of fragments, their path component must end in a trailing slash.

Currently supported schemes are "gs" for Google Cloud Storage, "s3" for Amazon S3, "azure" for Azure Cloud Storage, and "file" for a local file-system / NFS mount. Eg:

  • s3://bucket-name/a/sub-path/?profile=a-shared-credentials-profile
  • gs://bucket-name/a/sub-path/?
  • file:///a/local/volume/mount

FragmentStore implementations may support additional configuration which can be declared via URL query arguments. The meaning of these query arguments and values are specific to the store in question; consult FileStoreConfig, S3StoreConfig, and GSStoreConfig of the fragment package for details of available configuration.

func (FragmentStore) URL

func (fs FragmentStore) URL() *url.URL

URL returns the FragmentStore as a URL. The FragmentStore must Validate, or URL panics.

func (FragmentStore) Validate

func (fs FragmentStore) Validate() error

Validate returns an error if the FragmentStore is not well-formed.

type FragmentsRequest

type FragmentsRequest struct {
	// Header is attached by a proxying broker peer.
	Header *Header `protobuf:"bytes,1,opt,name=header,proto3" json:"header,omitempty"`
	// Journal to be read.
	Journal Journal `protobuf:"bytes,2,opt,name=journal,proto3,casttype=Journal" json:"journal,omitempty"`
	// BeginModTime is an optional field specifying an inclusive lower bound on
	// the modification timestamp for a fragment to be returned. The timestamp is
	// represented as seconds since the epoch.
	BeginModTime int64 `protobuf:"varint,3,opt,name=begin_mod_time,json=beginModTime,proto3" json:"begin_mod_time,omitempty"`
	// EndModTime is an optional field specifying an exclusive upper bound on
	// the modification timestamp for a fragment to be returned. The timestamp is
	// represented as seconds since the epoch.
	EndModTime int64 `protobuf:"varint,4,opt,name=end_mod_time,json=endModTime,proto3" json:"end_mod_time,omitempty"`
	// The NextPageToke value returned from a previous, continued
	// FragmentsRequest, if any.
	NextPageToken int64 `protobuf:"varint,5,opt,name=next_page_token,json=nextPageToken,proto3" json:"next_page_token,omitempty"`
	// PageLimit is an optional field specifying how many fragments to return
	// with the response. The default value for PageLimit is 1000.
	PageLimit int32 `protobuf:"varint,6,opt,name=page_limit,json=pageLimit,proto3" json:"page_limit,omitempty"`
	// SignatureTTL indicates that a temporary signed GET URL should be returned
	// with each response Fragment, valid for |signatureTTL|.
	SignatureTTL *time.Duration `protobuf:"bytes,7,opt,name=signatureTTL,proto3,stdduration" json:"signatureTTL,omitempty"`
	// If do_not_proxy is true, the broker will not proxy the request to another
	// broker on the client's behalf.
	DoNotProxy bool `protobuf:"varint,8,opt,name=do_not_proxy,json=doNotProxy,proto3" json:"do_not_proxy,omitempty"`
}

FragmentsRequest is the unary request message of the broker ListFragments RPC.

func (*FragmentsRequest) Descriptor

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

func (*FragmentsRequest) Equal added in v0.83.1

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

func (*FragmentsRequest) Marshal

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

func (*FragmentsRequest) MarshalTo

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

func (*FragmentsRequest) MarshalToSizedBuffer added in v0.86.1

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

func (*FragmentsRequest) ProtoMessage

func (*FragmentsRequest) ProtoMessage()

func (*FragmentsRequest) ProtoSize

func (m *FragmentsRequest) ProtoSize() (n int)

func (*FragmentsRequest) Reset

func (m *FragmentsRequest) Reset()

func (*FragmentsRequest) String

func (m *FragmentsRequest) String() string

func (*FragmentsRequest) Unmarshal

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

func (*FragmentsRequest) Validate

func (m *FragmentsRequest) Validate() error

func (*FragmentsRequest) XXX_DiscardUnknown

func (m *FragmentsRequest) XXX_DiscardUnknown()

func (*FragmentsRequest) XXX_Marshal

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

func (*FragmentsRequest) XXX_Merge

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

func (*FragmentsRequest) XXX_Size

func (m *FragmentsRequest) XXX_Size() int

func (*FragmentsRequest) XXX_Unmarshal

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

type FragmentsResponse

type FragmentsResponse struct {
	// Status of the Apply RPC.
	Status Status `protobuf:"varint,1,opt,name=status,proto3,enum=protocol.Status" json:"status,omitempty"`
	// Header of the response.
	Header    Header                        `protobuf:"bytes,2,opt,name=header,proto3" json:"header"`
	Fragments []FragmentsResponse__Fragment `protobuf:"bytes,3,rep,name=fragments,proto3" json:"fragments"`
	// The NextPageToke value to be returned on subsequent Fragments requests. If
	// the value is zero then there are no more fragments to be returned for this
	// page.
	NextPageToken int64 `protobuf:"varint,4,opt,name=next_page_token,json=nextPageToken,proto3" json:"next_page_token,omitempty"`
}

FragmentsResponse is the unary response message of the broker ListFragments RPC.

func (*FragmentsResponse) Descriptor

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

func (*FragmentsResponse) Equal added in v0.83.1

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

func (*FragmentsResponse) Marshal

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

func (*FragmentsResponse) MarshalTo

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

func (*FragmentsResponse) MarshalToSizedBuffer added in v0.86.1

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

func (*FragmentsResponse) ProtoMessage

func (*FragmentsResponse) ProtoMessage()

func (*FragmentsResponse) ProtoSize

func (m *FragmentsResponse) ProtoSize() (n int)

func (*FragmentsResponse) Reset

func (m *FragmentsResponse) Reset()

func (*FragmentsResponse) String

func (m *FragmentsResponse) String() string

func (*FragmentsResponse) Unmarshal

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

func (*FragmentsResponse) Validate

func (m *FragmentsResponse) Validate() error

func (*FragmentsResponse) XXX_DiscardUnknown

func (m *FragmentsResponse) XXX_DiscardUnknown()

func (*FragmentsResponse) XXX_Marshal

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

func (*FragmentsResponse) XXX_Merge

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

func (*FragmentsResponse) XXX_Size

func (m *FragmentsResponse) XXX_Size() int

func (*FragmentsResponse) XXX_Unmarshal

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

type FragmentsResponse__Fragment

type FragmentsResponse__Fragment struct {
	Spec Fragment `protobuf:"bytes,1,opt,name=spec,proto3" json:"spec"`
	// SignedURL is a temporary URL at which a direct GET of the Fragment may
	// be issued, signed by the broker's credentials. Set only if the request
	// specified a SignatureTTL.
	SignedUrl string `protobuf:"bytes,2,opt,name=signed_url,json=signedUrl,proto3" json:"signed_url,omitempty"`
}

Fragments of the Response.

func (*FragmentsResponse__Fragment) Descriptor

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

func (*FragmentsResponse__Fragment) Equal added in v0.83.1

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

func (*FragmentsResponse__Fragment) Marshal

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

func (*FragmentsResponse__Fragment) MarshalTo

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

func (*FragmentsResponse__Fragment) MarshalToSizedBuffer added in v0.86.1

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

func (*FragmentsResponse__Fragment) ProtoMessage

func (*FragmentsResponse__Fragment) ProtoMessage()

func (*FragmentsResponse__Fragment) ProtoSize

func (m *FragmentsResponse__Fragment) ProtoSize() (n int)

func (*FragmentsResponse__Fragment) Reset

func (m *FragmentsResponse__Fragment) Reset()

func (*FragmentsResponse__Fragment) String

func (m *FragmentsResponse__Fragment) String() string

func (*FragmentsResponse__Fragment) Unmarshal

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

func (*FragmentsResponse__Fragment) Validate

func (m *FragmentsResponse__Fragment) Validate() error

func (*FragmentsResponse__Fragment) XXX_DiscardUnknown

func (m *FragmentsResponse__Fragment) XXX_DiscardUnknown()

func (*FragmentsResponse__Fragment) XXX_Marshal

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

func (*FragmentsResponse__Fragment) XXX_Merge

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

func (*FragmentsResponse__Fragment) XXX_Size

func (m *FragmentsResponse__Fragment) XXX_Size() int

func (*FragmentsResponse__Fragment) XXX_Unmarshal

func (m *FragmentsResponse__Fragment) XXX_Unmarshal(b []byte) error
type Header struct {
	// ID of the process responsible for request processing. May be empty iff
	// Header is being used within a proxied request, and that request may be
	// dispatched to any member of the Route.
	ProcessId ProcessSpec_ID `protobuf:"bytes,1,opt,name=process_id,json=processId,proto3" json:"process_id"`
	// Route of processes specifically responsible for this RPC, or an empty Route
	// if any process is capable of serving the RPC.
	Route Route       `protobuf:"bytes,2,opt,name=route,proto3" json:"route"`
	Etcd  Header_Etcd `protobuf:"bytes,3,opt,name=etcd,proto3" json:"etcd"`
}

Header captures metadata such as the process responsible for processing an RPC, and its effective Etcd state.

func (*Header) Descriptor

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

func (*Header) Equal added in v0.83.1

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

func (*Header) Marshal

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

func (*Header) MarshalTo

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

func (*Header) MarshalToSizedBuffer added in v0.86.1

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

func (*Header) ProtoMessage

func (*Header) ProtoMessage()

func (*Header) ProtoSize

func (m *Header) ProtoSize() (n int)

func (*Header) Reset

func (m *Header) Reset()

func (*Header) String

func (m *Header) String() string

func (*Header) Unmarshal

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

func (Header) Validate

func (m Header) Validate() error

Validate returns an error if the Header is not well-formed.

func (*Header) XXX_DiscardUnknown

func (m *Header) XXX_DiscardUnknown()

func (*Header) XXX_Marshal

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

func (*Header) XXX_Merge

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

func (*Header) XXX_Size

func (m *Header) XXX_Size() int

func (*Header) XXX_Unmarshal

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

type Header_Etcd

type Header_Etcd struct {
	// cluster_id is the ID of the cluster.
	ClusterId uint64 `protobuf:"varint,1,opt,name=cluster_id,json=clusterId,proto3" json:"cluster_id,omitempty"`
	// member_id is the ID of the member.
	MemberId uint64 `protobuf:"varint,2,opt,name=member_id,json=memberId,proto3" json:"member_id,omitempty"`
	// revision is the Etcd key-value store revision when the request was
	// applied.
	Revision int64 `protobuf:"varint,3,opt,name=revision,proto3" json:"revision,omitempty"`
	// raft_term is the raft term when the request was applied.
	RaftTerm uint64 `protobuf:"varint,4,opt,name=raft_term,json=raftTerm,proto3" json:"raft_term,omitempty"`
}

Etcd represents the effective Etcd MVCC state under which a Gazette broker is operating in its processing of requests and responses. Its inclusion allows brokers to reason about relative "happened before" Revision ordering of apparent routing conflicts in proxied or replicated requests, as well as enabling sanity checks over equality of Etcd ClusterId (and precluding, for example, split-brain scenarios where different brokers are backed by different Etcd clusters). Etcd is kept in sync with etcdserverpb.ResponseHeader.

func (*Header_Etcd) Descriptor

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

func (*Header_Etcd) Equal added in v0.83.1

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

func (*Header_Etcd) Marshal

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

func (*Header_Etcd) MarshalTo

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

func (*Header_Etcd) MarshalToSizedBuffer added in v0.86.1

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

func (*Header_Etcd) ProtoMessage

func (*Header_Etcd) ProtoMessage()

func (*Header_Etcd) ProtoSize

func (m *Header_Etcd) ProtoSize() (n int)

func (*Header_Etcd) Reset

func (m *Header_Etcd) Reset()

func (*Header_Etcd) String

func (m *Header_Etcd) String() string

func (*Header_Etcd) Unmarshal

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

func (Header_Etcd) Validate

func (m Header_Etcd) Validate() error

Validate returns an error if the Header_Etcd is not well-formed.

func (*Header_Etcd) XXX_DiscardUnknown

func (m *Header_Etcd) XXX_DiscardUnknown()

func (*Header_Etcd) XXX_Marshal

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

func (*Header_Etcd) XXX_Merge

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

func (*Header_Etcd) XXX_Size

func (m *Header_Etcd) XXX_Size() int

func (*Header_Etcd) XXX_Unmarshal

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

type Journal

type Journal string

Journal uniquely identifies a journal brokered by Gazette. By convention, journals are named using a forward-slash notation which captures their hierarchical relationships into organizations, topics and partitions. For example, a Journal might be: "company-journals/interesting-topic/part-1234"

func (Journal) SplitMeta added in v0.89.0

func (n Journal) SplitMeta() (Journal, string)

SplitMeta splits off a ";meta" path segment of this Journal, separately returning the Journal and the ";meta" remainder (including leading ';' delimiter). If there is no metadata segment, the Journal is returned with an empty string.

func (Journal) String

func (n Journal) String() string

String returns the Journal as a string.

func (Journal) StripMeta added in v0.89.0

func (n Journal) StripMeta() Journal

StripMeta returns this Journal with a ";meta" suffix removed.

func (Journal) Validate

func (n Journal) Validate() error

Validate returns an error if the Journal is not well-formed. It must draw from a restricted set of allowed path runes, be a clean path (as defined by path.Clean), and must not begin with a '/'. A Journal metadata component, if present, must similarly consist only of allowed token runes.

type JournalClient

type JournalClient interface {
	// List Journals, their JournalSpecs and current Routes.
	List(ctx context.Context, in *ListRequest, opts ...grpc.CallOption) (*ListResponse, error)
	// Apply changes to the collection of Journals managed by the brokers.
	Apply(ctx context.Context, in *ApplyRequest, opts ...grpc.CallOption) (*ApplyResponse, error)
	// Read from a specific Journal.
	Read(ctx context.Context, in *ReadRequest, opts ...grpc.CallOption) (Journal_ReadClient, error)
	// Append content to a specific Journal.
	Append(ctx context.Context, opts ...grpc.CallOption) (Journal_AppendClient, error)
	// Replicate appended content of a Journal. Replicate is used between broker
	// peers in the course of processing Append transactions, but is not intended
	// for direct use by clients.
	Replicate(ctx context.Context, opts ...grpc.CallOption) (Journal_ReplicateClient, error)
	// List Fragments of a Journal.
	ListFragments(ctx context.Context, in *FragmentsRequest, opts ...grpc.CallOption) (*FragmentsResponse, error)
}

JournalClient is the client API for Journal service.

For semantics around ctx use and closing/ending streaming RPCs, please refer to https://godoc.org/google.golang.org/grpc#ClientConn.NewStream.

func NewJournalClient

func NewJournalClient(cc *grpc.ClientConn) JournalClient

type JournalServer

type JournalServer interface {
	// List Journals, their JournalSpecs and current Routes.
	List(context.Context, *ListRequest) (*ListResponse, error)
	// Apply changes to the collection of Journals managed by the brokers.
	Apply(context.Context, *ApplyRequest) (*ApplyResponse, error)
	// Read from a specific Journal.
	Read(*ReadRequest, Journal_ReadServer) error
	// Append content to a specific Journal.
	Append(Journal_AppendServer) error
	// Replicate appended content of a Journal. Replicate is used between broker
	// peers in the course of processing Append transactions, but is not intended
	// for direct use by clients.
	Replicate(Journal_ReplicateServer) error
	// List Fragments of a Journal.
	ListFragments(context.Context, *FragmentsRequest) (*FragmentsResponse, error)
}

JournalServer is the server API for Journal service.

type JournalSpec

type JournalSpec struct {
	// Name of the Journal.
	Name Journal `protobuf:"bytes,1,opt,name=name,proto3,casttype=Journal" json:"name,omitempty" yaml:",omitempty"`
	// Desired replication of this Journal. This defines the Journal's tolerance
	// to broker failures before data loss can occur (eg, a replication factor
	// of three means two failures are tolerated).
	Replication int32 `protobuf:"varint,2,opt,name=replication,proto3" json:"replication,omitempty" yaml:",omitempty"`
	// User-defined Labels of this JournalSpec. Two label names are reserved
	// and may not be used within a JournalSpec's Labels: "name" and "prefix".
	LabelSet `protobuf:"bytes,3,opt,name=labels,proto3,embedded=labels" json:"labels" yaml:",omitempty,inline"`
	Fragment JournalSpec_Fragment `protobuf:"bytes,4,opt,name=fragment,proto3" json:"fragment" yaml:",omitempty"`
	// Flags of the Journal, as a combination of Flag enum values. The Flag enum
	// is not used directly, as protobuf enums do not allow for or'ed bitfields.
	Flags JournalSpec_Flag `protobuf:"varint,6,opt,name=flags,proto3,casttype=JournalSpec_Flag" json:"flags,omitempty" yaml:",omitempty"`
	// Maximum rate, in bytes-per-second, at which appends of this journal will
	// be processed. If zero (the default), no rate limiting is applied. A global
	// rate limit still may be in effect, in which case the effective rate is the
	// smaller of the journal vs global rate.
	MaxAppendRate int64 `` /* 136-byte string literal not displayed */
}

JournalSpec describes a Journal and its configuration.

func IntersectJournalSpecs

func IntersectJournalSpecs(a, b JournalSpec) JournalSpec

IntersectJournalSpecs returns a JournalSpec having a non-zero-valued field for each field value which is shared between |a| and |b|.

func SubtractJournalSpecs

func SubtractJournalSpecs(a, b JournalSpec) JournalSpec

SubtractJournalSpecs returns a JournalSpec derived from |a| but having a zero-valued field for each field which is matched by |b|.

func UnionJournalSpecs

func UnionJournalSpecs(a, b JournalSpec) JournalSpec

UnionJournalSpecs returns a JournalSpec combining all non-zero-valued fields across |a| and |b|. Where both |a| and |b| provide a non-zero value for a field, the value of |a| is retained.

func (*JournalSpec) Descriptor

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

func (*JournalSpec) DesiredReplication

func (m *JournalSpec) DesiredReplication() int

DesiredReplication returns the configured Replication of the spec. It implements allocator.ItemValue.

func (*JournalSpec) Equal added in v0.83.1

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

func (*JournalSpec) Marshal

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

func (*JournalSpec) MarshalString

func (m *JournalSpec) MarshalString() string

MarshalString returns the marshaled encoding of the JournalSpec as a string.

func (*JournalSpec) MarshalTo

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

func (*JournalSpec) MarshalToSizedBuffer added in v0.86.1

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

func (*JournalSpec) ProtoMessage

func (*JournalSpec) ProtoMessage()

func (*JournalSpec) ProtoSize

func (m *JournalSpec) ProtoSize() (n int)

func (*JournalSpec) Reset

func (m *JournalSpec) Reset()

func (*JournalSpec) String

func (m *JournalSpec) String() string

func (*JournalSpec) Unmarshal

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

func (*JournalSpec) Validate

func (m *JournalSpec) Validate() error

Validate returns an error if the JournalSpec is not well-formed.

func (*JournalSpec) XXX_DiscardUnknown

func (m *JournalSpec) XXX_DiscardUnknown()

func (*JournalSpec) XXX_Marshal

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

func (*JournalSpec) XXX_Merge

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

func (*JournalSpec) XXX_Size

func (m *JournalSpec) XXX_Size() int

func (*JournalSpec) XXX_Unmarshal

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

type JournalSpec_Flag

type JournalSpec_Flag int32

Flags define Journal IO control behaviors. Where possible, flags are named after an equivalent POSIX flag.

const (
	// NOT_SPECIFIED is considered as equivalent to O_RDWR by the broker. When
	// JournalSpecs are union'ed (eg, by the `journalspace` pkg), NOT_SPECIFIED
	// is considered as unset relative to any other non-zero Flag value.
	JournalSpec_NOT_SPECIFIED JournalSpec_Flag = 0
	// The Journal is available for reads (only).
	JournalSpec_O_RDONLY JournalSpec_Flag = 1
	// The Journal is available for writes (only).
	JournalSpec_O_WRONLY JournalSpec_Flag = 2
	// The Journal may be used for reads or writes.
	JournalSpec_O_RDWR JournalSpec_Flag = 4
)

func (JournalSpec_Flag) EnumDescriptor

func (JournalSpec_Flag) EnumDescriptor() ([]byte, []int)

func (JournalSpec_Flag) MarshalYAML

func (x JournalSpec_Flag) MarshalYAML() (interface{}, error)

MarshalYAML maps the JournalSpec_Flag to a YAML value.

func (JournalSpec_Flag) MayRead

func (x JournalSpec_Flag) MayRead() bool

MayRead returns whether reads are permitted.

func (JournalSpec_Flag) MayWrite

func (x JournalSpec_Flag) MayWrite() bool

MayWrite returns whether writes are permitted.

func (JournalSpec_Flag) String

func (x JournalSpec_Flag) String() string

func (*JournalSpec_Flag) UnmarshalYAML

func (x *JournalSpec_Flag) UnmarshalYAML(unmarshal func(interface{}) error) error

UnmarshalYAML maps a YAML integer directly to a Flag value, or a YAML string to a Flag with corresponding enum name.

func (JournalSpec_Flag) Validate

func (x JournalSpec_Flag) Validate() error

Validate returns an error if the JournalSpec_Flag is malformed.

type JournalSpec_Fragment

type JournalSpec_Fragment struct {
	// Target content length of each Fragment. In normal operation after
	// Fragments reach at least this length, they will be closed and new ones
	// begun. Note lengths may be smaller at times (eg, due to changes in
	// Journal routing topology). Content length differs from Fragment file
	// size, in that the former reflects uncompressed bytes.
	Length int64 `protobuf:"varint,1,opt,name=length,proto3" json:"length,omitempty" yaml:",omitempty"`
	// Codec used to compress Journal Fragments.
	CompressionCodec CompressionCodec `` /* 176-byte string literal not displayed */
	// Storage backend base path for this Journal's Fragments. Must be in URL
	// form, with the choice of backend defined by the scheme. The full path of
	// a Journal's Fragment is derived by joining the store path with the
	// Fragment's ContentPath. Eg, given a fragment_store of
	//   "s3://My-AWS-bucket/a/prefix" and a JournalSpec of name "my/journal",
	// a complete Fragment path might be:
	//   "s3://My-AWS-bucket/a/prefix/my/journal/000123-000456-789abcdef.gzip
	//
	// Multiple stores may be specified, in which case the Journal's Fragments
	// are the union of all Fragments present across all stores, and new
	// Fragments always persist to the first specified store. This can be
	// helpful in performing incremental migrations, where new Journal content
	// is written to the new store, while content in the old store remains
	// available (and, depending on fragment_retention or recovery log pruning,
	// may eventually be removed).
	//
	// If no stores are specified, the Journal is still use-able but will
	// not persist Fragments to any a backing fragment store. This allows for
	// real-time streaming use cases where reads of historical data are not
	// needed.
	Stores []FragmentStore `protobuf:"bytes,3,rep,name=stores,proto3,casttype=FragmentStore" json:"stores,omitempty" yaml:",omitempty"`
	// Interval of time between refreshes of remote Fragment listings from
	// configured fragment_stores.
	RefreshInterval time.Duration `` /* 142-byte string literal not displayed */
	// Retention duration for historical Fragments of this Journal within the
	// Fragment stores. If less than or equal to zero, Fragments are retained
	// indefinitely.
	Retention time.Duration `protobuf:"bytes,5,opt,name=retention,proto3,stdduration" json:"retention" yaml:",omitempty"`
	// Flush interval defines a uniform UTC time segment which, when passed,
	// will prompt brokers to close and persist a fragment presently being
	// written.
	//
	// Flush interval may be helpful in integrating the journal with a regularly
	// scheduled batch work-flow which processes new files from the fragment
	// store and has no particular awareness of Gazette. For example, setting
	// flush_interval to 3600s will cause brokers to persist their present
	// fragment on the hour, every hour, even if it has not yet reached its
	// target length. A batch work-flow running at 5 minutes past the hour is
	// then reasonably assured of seeing all events from the past hour.
	//
	// See also "gazctl journals fragments --help" for more discussion.
	FlushInterval time.Duration `` /* 134-byte string literal not displayed */
	// Path postfix template is a Go template which evaluates to a partial
	// path under which fragments are persisted to the store. A complete
	// fragment path is constructed by appending path components from the
	// fragment store, then the journal name, and then the postfix template.
	// Path post-fixes can help in maintaining Hive compatible partitioning
	// over fragment creation time. The fields ".Spool" and ".JournalSpec"
	// are available for introspection in the template. For example,
	// to partition on the UTC date and hour of creation, use:
	//
	//    date={{ .Spool.FirstAppendTime.Format "2006-01-02" }}/hour={{
	//    .Spool.FirstAppendTime.Format "15" }}
	//
	// Which will produce a path postfix like "date=2019-11-19/hour=22".
	PathPostfixTemplate string `` /* 159-byte string literal not displayed */
}

Fragment is JournalSpec configuration which pertains to the creation, persistence, and indexing of the Journal's Fragments.

func (*JournalSpec_Fragment) Descriptor

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

func (*JournalSpec_Fragment) Equal added in v0.83.1

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

func (*JournalSpec_Fragment) Marshal

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

func (*JournalSpec_Fragment) MarshalTo

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

func (*JournalSpec_Fragment) MarshalToSizedBuffer added in v0.86.1

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

func (*JournalSpec_Fragment) ProtoMessage

func (*JournalSpec_Fragment) ProtoMessage()

func (*JournalSpec_Fragment) ProtoSize

func (m *JournalSpec_Fragment) ProtoSize() (n int)

func (*JournalSpec_Fragment) Reset

func (m *JournalSpec_Fragment) Reset()

func (*JournalSpec_Fragment) String

func (m *JournalSpec_Fragment) String() string

func (*JournalSpec_Fragment) Unmarshal

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

func (*JournalSpec_Fragment) Validate

func (m *JournalSpec_Fragment) Validate() error

Validate returns an error if the JournalSpec_Fragment is not well-formed.

func (*JournalSpec_Fragment) XXX_DiscardUnknown

func (m *JournalSpec_Fragment) XXX_DiscardUnknown()

func (*JournalSpec_Fragment) XXX_Marshal

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

func (*JournalSpec_Fragment) XXX_Merge

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

func (*JournalSpec_Fragment) XXX_Size

func (m *JournalSpec_Fragment) XXX_Size() int

func (*JournalSpec_Fragment) XXX_Unmarshal

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

type Journal_AppendClient

type Journal_AppendClient interface {
	Send(*AppendRequest) error
	CloseAndRecv() (*AppendResponse, error)
	grpc.ClientStream
}

type Journal_AppendServer

type Journal_AppendServer interface {
	SendAndClose(*AppendResponse) error
	Recv() (*AppendRequest, error)
	grpc.ServerStream
}

type Journal_ReadClient

type Journal_ReadClient interface {
	Recv() (*ReadResponse, error)
	grpc.ClientStream
}

type Journal_ReadServer

type Journal_ReadServer interface {
	Send(*ReadResponse) error
	grpc.ServerStream
}

type Journal_ReplicateClient

type Journal_ReplicateClient interface {
	Send(*ReplicateRequest) error
	Recv() (*ReplicateResponse, error)
	grpc.ClientStream
}

type Journal_ReplicateServer

type Journal_ReplicateServer interface {
	Send(*ReplicateResponse) error
	Recv() (*ReplicateRequest, error)
	grpc.ServerStream
}

type Label

type Label struct {
	Name  string `protobuf:"bytes,1,opt,name=name,proto3" json:"name,omitempty"`
	Value string `protobuf:"bytes,2,opt,name=value,proto3" json:"value,omitempty"`
}

Label defines a key & value pair which can be attached to entities like JournalSpecs and BrokerSpecs. Labels may be used to provide identifying attributes which do not directly imply semantics to the core system, but are meaningful to users or for higher-level Gazette tools.

func (*Label) Descriptor

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

func (*Label) Equal added in v0.83.1

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

func (*Label) Marshal

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

func (*Label) MarshalTo

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

func (*Label) MarshalToSizedBuffer added in v0.86.1

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

func (*Label) ProtoMessage

func (*Label) ProtoMessage()

func (*Label) ProtoSize

func (m *Label) ProtoSize() (n int)

func (*Label) Reset

func (m *Label) Reset()

func (*Label) String

func (m *Label) String() string

func (*Label) Unmarshal

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

func (Label) Validate

func (m Label) Validate() error

Validate returns an error if the Label is not well-formed.

func (*Label) XXX_DiscardUnknown

func (m *Label) XXX_DiscardUnknown()

func (*Label) XXX_Marshal

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

func (*Label) XXX_Merge

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

func (*Label) XXX_Size

func (m *Label) XXX_Size() int

func (*Label) XXX_Unmarshal

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

type LabelSelector

type LabelSelector struct {
	// Include is Labels which must be matched for a LabelSet to be selected. If
	// empty, all Labels are included. An include Label with empty ("") value is
	// matched by a Label of the same name having any value.
	Include LabelSet `protobuf:"bytes,1,opt,name=include,proto3" json:"include"`
	// Exclude is Labels which cannot be matched for a LabelSet to be selected. If
	// empty, no Labels are excluded. An exclude Label with empty ("") value
	// excludes a Label of the same name having any value.
	Exclude LabelSet `protobuf:"bytes,2,opt,name=exclude,proto3" json:"exclude"`
}

LabelSelector defines a filter over LabelSets.

func ParseLabelSelector

func ParseLabelSelector(s string) (LabelSelector, error)

ParseLabelSelector parses a LabelSelector string. Selector strings are composed of a comma-separate list of selector expressions. Allowed expression types are equality, in-equality, set membership, set exclusion, existence, and non-existence. Eg:

  • "foo = bar" requires that label "foo" be present with value "bar"
  • "foo != bar" requires that label "foo" not be present with value "bar"
  • "foo" requires that label "foo" be present (with any value).
  • "!foo" requires that label "foo" not be present.
  • "foo in (bar,baz)" requires that "foo" be present with either "bar" or "baz".
  • "foo notin (bar,baz)" requires that "foo", if present, not have value "bar" or "baz".

Additional examples of composite expressions:

  • "topic in (topic/one, topic/two), prefix=/my/journal/prefix"
  • "env in (production, qa), tier not in (frontend,backend), partition"

ParseLabelSelector is invariant to _reasonable_ spacing: eg, "not in" and "notin" may be used interchangeably, as may "==" and "=", with or without single spacing on either side.

func (*LabelSelector) Descriptor

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

func (*LabelSelector) Equal added in v0.83.1

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

func (*LabelSelector) Marshal

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

func (*LabelSelector) MarshalTo

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

func (*LabelSelector) MarshalToSizedBuffer added in v0.86.1

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

func (LabelSelector) Matches

func (m LabelSelector) Matches(s LabelSet) bool

Matches returns whether the LabelSet is matched by the LabelSelector.

func (*LabelSelector) ProtoMessage

func (*LabelSelector) ProtoMessage()

func (*LabelSelector) ProtoSize

func (m *LabelSelector) ProtoSize() (n int)

func (*LabelSelector) Reset

func (m *LabelSelector) Reset()

func (LabelSelector) String

func (s LabelSelector) String() string

String returns a canonical string representation of the LabelSelector.

func (*LabelSelector) Unmarshal

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

func (LabelSelector) Validate

func (m LabelSelector) Validate() error

Validate returns an error if the LabelSelector is not well-formed.

func (*LabelSelector) XXX_DiscardUnknown

func (m *LabelSelector) XXX_DiscardUnknown()

func (*LabelSelector) XXX_Marshal

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

func (*LabelSelector) XXX_Merge

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

func (*LabelSelector) XXX_Size

func (m *LabelSelector) XXX_Size() int

func (*LabelSelector) XXX_Unmarshal

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

type LabelSet

type LabelSet struct {
	// Labels of the set. Instances must be unique and sorted over (Name, Value).
	Labels []Label `protobuf:"bytes,1,rep,name=labels,proto3" json:"labels" yaml:",omitempty"`
}

LabelSet is a collection of labels and their values.

func ExtractJournalSpecMetaLabels

func ExtractJournalSpecMetaLabels(spec *JournalSpec, out LabelSet) LabelSet

ExtractJournalSpecMetaLabels adds to the LabelSet a singular label "name", with value of the JournalSpec Name, and multi-label "prefix", having a value for each path component prefix of Name.

func IntersectLabelSets

func IntersectLabelSets(lhs, rhs, out LabelSet) LabelSet

IntersectLabelSets returns the LabelSet having all labels present in both |lhs| and |rhs| with matched values.

func MustLabelSet

func MustLabelSet(nv ...string) (set LabelSet)

MustLabelSet is a convenience for constructing LabelSets from a sequence of Name, Value arguments. The result LabelSet must Validate or MustLabelSet panics.

func SubtractLabelSet

func SubtractLabelSet(lhs, rhs, out LabelSet) LabelSet

SubtractLabelSets returns the LabelSet having labels in |lhs| which are not present in |rhs| with matched values.

func UnionLabelSets

func UnionLabelSets(lhs, rhs, out LabelSet) LabelSet

UnionLabelSets returns the LabelSet having all labels present in either |lhs| or |rhs|. Where both |lhs| and |rhs| have values for a label, those of |lhs| are preferred.

func (*LabelSet) AddValue

func (m *LabelSet) AddValue(name, value string)

AddValue adds Label |name| with |value|, retaining any existing Labels |name|.

func (*LabelSet) Assign added in v0.83.1

func (m *LabelSet) Assign(other *LabelSet)

Assign the labels of the |other| set to this LabelSet. If |other| is nil, this LabelSet is emptied.

func (*LabelSet) Descriptor

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

func (*LabelSet) Equal added in v0.83.1

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

func (*LabelSet) Marshal

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

func (*LabelSet) MarshalTo

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

func (*LabelSet) MarshalToSizedBuffer added in v0.86.1

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

func (*LabelSet) ProtoMessage

func (*LabelSet) ProtoMessage()

func (*LabelSet) ProtoSize

func (m *LabelSet) ProtoSize() (n int)

func (*LabelSet) Remove

func (m *LabelSet) Remove(name string)

Remove all instances of Labels |name|.

func (*LabelSet) Reset

func (m *LabelSet) Reset()

func (*LabelSet) SetValue

func (m *LabelSet) SetValue(name, value string)

SetValue adds Label |name| with |value|, replacing any existing Labels |name|.

func (*LabelSet) String

func (m *LabelSet) String() string

func (*LabelSet) Unmarshal

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

func (LabelSet) Validate

func (m LabelSet) Validate() error

Validate returns an error if the LabelSet is not well-formed.

func (LabelSet) ValueOf

func (m LabelSet) ValueOf(name string) string

ValueOf returns the first value of Label |name|, or "" if it doesn't exist in the LabelSet.

func (LabelSet) ValuesOf

func (m LabelSet) ValuesOf(name string) (values []string)

ValuesOf returns the values of Label |name|, or nil if it doesn't exist in the LabelSet.

func (*LabelSet) XXX_DiscardUnknown

func (m *LabelSet) XXX_DiscardUnknown()

func (*LabelSet) XXX_Marshal

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

func (*LabelSet) XXX_Merge

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

func (*LabelSet) XXX_Size

func (m *LabelSet) XXX_Size() int

func (*LabelSet) XXX_Unmarshal

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

type ListRequest

type ListRequest struct {
	// Selector optionally refines the set of journals which will be enumerated.
	// If zero-valued, all journals are returned. Otherwise, only JournalSpecs
	// matching the LabelSelector will be returned. Two meta-labels "name" and
	// "prefix" are additionally supported by the selector, where:
	//   * name=examples/a-name will match a JournalSpec with Name
	//   "examples/a-name"
	//   * prefix=examples/ will match any JournalSpec having prefix "examples/".
	//     The prefix Label value must end in '/'.
	Selector LabelSelector `protobuf:"bytes,1,opt,name=selector,proto3" json:"selector"`
}

ListRequest is the unary request message of the broker List RPC.

func (*ListRequest) Descriptor

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

func (*ListRequest) Equal added in v0.83.1

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

func (*ListRequest) Marshal

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

func (*ListRequest) MarshalTo

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

func (*ListRequest) MarshalToSizedBuffer added in v0.86.1

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

func (*ListRequest) ProtoMessage

func (*ListRequest) ProtoMessage()

func (*ListRequest) ProtoSize

func (m *ListRequest) ProtoSize() (n int)

func (*ListRequest) Reset

func (m *ListRequest) Reset()

func (*ListRequest) String

func (m *ListRequest) String() string

func (*ListRequest) Unmarshal

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

func (*ListRequest) Validate

func (m *ListRequest) Validate() error

func (*ListRequest) XXX_DiscardUnknown

func (m *ListRequest) XXX_DiscardUnknown()

func (*ListRequest) XXX_Marshal

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

func (*ListRequest) XXX_Merge

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

func (*ListRequest) XXX_Size

func (m *ListRequest) XXX_Size() int

func (*ListRequest) XXX_Unmarshal

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

type ListResponse

type ListResponse struct {
	// Status of the List RPC.
	Status Status `protobuf:"varint,1,opt,name=status,proto3,enum=protocol.Status" json:"status,omitempty"`
	// Header of the response.
	Header   Header                 `protobuf:"bytes,2,opt,name=header,proto3" json:"header"`
	Journals []ListResponse_Journal `protobuf:"bytes,3,rep,name=journals,proto3" json:"journals"`
}

ListResponse is the unary response message of the broker List RPC.

func (*ListResponse) Descriptor

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

func (*ListResponse) Equal added in v0.83.1

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

func (*ListResponse) Marshal

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

func (*ListResponse) MarshalTo

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

func (*ListResponse) MarshalToSizedBuffer added in v0.86.1

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

func (*ListResponse) ProtoMessage

func (*ListResponse) ProtoMessage()

func (*ListResponse) ProtoSize

func (m *ListResponse) ProtoSize() (n int)

func (*ListResponse) Reset

func (m *ListResponse) Reset()

func (*ListResponse) String

func (m *ListResponse) String() string

func (*ListResponse) Unmarshal

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

func (*ListResponse) Validate

func (m *ListResponse) Validate() error

func (*ListResponse) XXX_DiscardUnknown

func (m *ListResponse) XXX_DiscardUnknown()

func (*ListResponse) XXX_Marshal

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

func (*ListResponse) XXX_Merge

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

func (*ListResponse) XXX_Size

func (m *ListResponse) XXX_Size() int

func (*ListResponse) XXX_Unmarshal

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

type ListResponse_Journal

type ListResponse_Journal struct {
	Spec JournalSpec `protobuf:"bytes,1,opt,name=spec,proto3" json:"spec"`
	// Current ModRevision of the JournalSpec.
	ModRevision int64 `protobuf:"varint,2,opt,name=mod_revision,json=modRevision,proto3" json:"mod_revision,omitempty"`
	// Route of the journal, including endpoints.
	Route Route `protobuf:"bytes,3,opt,name=route,proto3" json:"route"`
}

Journals of the response.

func (*ListResponse_Journal) Descriptor

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

func (*ListResponse_Journal) Equal added in v0.83.1

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

func (*ListResponse_Journal) Marshal

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

func (*ListResponse_Journal) MarshalTo

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

func (*ListResponse_Journal) MarshalToSizedBuffer added in v0.86.1

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

func (*ListResponse_Journal) ProtoMessage

func (*ListResponse_Journal) ProtoMessage()

func (*ListResponse_Journal) ProtoSize

func (m *ListResponse_Journal) ProtoSize() (n int)

func (*ListResponse_Journal) Reset

func (m *ListResponse_Journal) Reset()

func (*ListResponse_Journal) String

func (m *ListResponse_Journal) String() string

func (*ListResponse_Journal) Unmarshal

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

func (*ListResponse_Journal) Validate

func (m *ListResponse_Journal) Validate() error

func (*ListResponse_Journal) XXX_DiscardUnknown

func (m *ListResponse_Journal) XXX_DiscardUnknown()

func (*ListResponse_Journal) XXX_Marshal

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

func (*ListResponse_Journal) XXX_Merge

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

func (*ListResponse_Journal) XXX_Size

func (m *ListResponse_Journal) XXX_Size() int

func (*ListResponse_Journal) XXX_Unmarshal

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

type NoopDispatchRouter

type NoopDispatchRouter struct{}

NoopDispatchRouter is a DispatchRouter which doesn't route.

func (NoopDispatchRouter) IsNoopRouter

func (NoopDispatchRouter) IsNoopRouter() bool

func (NoopDispatchRouter) Route

func (NoopDispatchRouter) UpdateRoute

func (NoopDispatchRouter) UpdateRoute(string, *Route)

type Offset added in v0.83.1

type Offset = int64

Offset is a byte offset of a Journal.

type Offsets added in v0.83.1

type Offsets map[Journal]Offset

Offsets is a map of Journals and Offsets.

func (Offsets) Copy added in v0.83.1

func (o Offsets) Copy() Offsets

Copy allocates and returns a copy of Offsets.

func (Offsets) Validate added in v0.83.1

func (o Offsets) Validate() error

Validate returns an error if the Offsets are not well-formed.

type ProcessSpec

type ProcessSpec struct {
	Id ProcessSpec_ID `protobuf:"bytes,1,opt,name=id,proto3" json:"id"`
	// Advertised URL of the process.
	Endpoint Endpoint `protobuf:"bytes,2,opt,name=endpoint,proto3,casttype=Endpoint" json:"endpoint,omitempty"`
}

ProcessSpec describes a uniquely identified process and its addressable endpoint.

func (*ProcessSpec) Descriptor

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

func (*ProcessSpec) Equal added in v0.83.1

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

func (*ProcessSpec) GetEndpoint

func (m *ProcessSpec) GetEndpoint() Endpoint

func (*ProcessSpec) GetId

func (m *ProcessSpec) GetId() ProcessSpec_ID

func (*ProcessSpec) Marshal

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

func (*ProcessSpec) MarshalTo

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

func (*ProcessSpec) MarshalToSizedBuffer added in v0.86.1

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

func (*ProcessSpec) ProtoMessage

func (*ProcessSpec) ProtoMessage()

func (*ProcessSpec) ProtoSize

func (m *ProcessSpec) ProtoSize() (n int)

func (*ProcessSpec) Reset

func (m *ProcessSpec) Reset()

func (*ProcessSpec) String

func (m *ProcessSpec) String() string

func (*ProcessSpec) Unmarshal

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

func (*ProcessSpec) Validate

func (m *ProcessSpec) Validate() error

Validate returns an error if the ProcessSpec is not well-formed.

func (*ProcessSpec) XXX_DiscardUnknown

func (m *ProcessSpec) XXX_DiscardUnknown()

func (*ProcessSpec) XXX_Marshal

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

func (*ProcessSpec) XXX_Merge

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

func (*ProcessSpec) XXX_Size

func (m *ProcessSpec) XXX_Size() int

func (*ProcessSpec) XXX_Unmarshal

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

type ProcessSpec_ID

type ProcessSpec_ID struct {
	// "Zone" in which the process is running. Zones may be AWS, Azure, or
	// Google Cloud Platform zone identifiers, or rack locations within a colo,
	// or given some other custom meaning. Gazette will replicate across
	// multiple zones, and seeks to minimize traffic which must cross zones (for
	// example, by proxying reads to a broker in the current zone).
	Zone string `protobuf:"bytes,1,opt,name=zone,proto3" json:"zone,omitempty"`
	// Unique suffix of the process within |zone|. It is permissible for a
	// suffix value to repeat across zones, but never within zones. In practice,
	// it's recommended to use a FQDN, Kubernetes Pod name, or comparable unique
	// and self-describing value as the ID suffix.
	Suffix string `protobuf:"bytes,2,opt,name=suffix,proto3" json:"suffix,omitempty"`
}

ID composes a zone and a suffix to uniquely identify a ProcessSpec.

func (*ProcessSpec_ID) Descriptor

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

func (*ProcessSpec_ID) Equal added in v0.83.1

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

func (ProcessSpec_ID) Less

func (m ProcessSpec_ID) Less(other ProcessSpec_ID) bool

Less returns whether the ProcessSpec_ID is less than the argument BrokerSpec_ID, under (Zone, Suffix) ordering.

func (*ProcessSpec_ID) Marshal

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

func (*ProcessSpec_ID) MarshalTo

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

func (*ProcessSpec_ID) MarshalToSizedBuffer added in v0.86.1

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

func (*ProcessSpec_ID) ProtoMessage

func (*ProcessSpec_ID) ProtoMessage()

func (*ProcessSpec_ID) ProtoSize

func (m *ProcessSpec_ID) ProtoSize() (n int)

func (*ProcessSpec_ID) Reset

func (m *ProcessSpec_ID) Reset()

func (*ProcessSpec_ID) String

func (m *ProcessSpec_ID) String() string

func (*ProcessSpec_ID) Unmarshal

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

func (ProcessSpec_ID) Validate

func (m ProcessSpec_ID) Validate() error

Validate returns an error if the ProcessSpec_ID is not well-formed.

func (*ProcessSpec_ID) XXX_DiscardUnknown

func (m *ProcessSpec_ID) XXX_DiscardUnknown()

func (*ProcessSpec_ID) XXX_Marshal

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

func (*ProcessSpec_ID) XXX_Merge

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

func (*ProcessSpec_ID) XXX_Size

func (m *ProcessSpec_ID) XXX_Size() int

func (*ProcessSpec_ID) XXX_Unmarshal

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

type ReadRequest

type ReadRequest struct {
	// Header is attached by a proxying broker peer.
	Header *Header `protobuf:"bytes,1,opt,name=header,proto3" json:"header,omitempty"`
	// Journal to be read.
	Journal Journal `protobuf:"bytes,2,opt,name=journal,proto3,casttype=Journal" json:"journal,omitempty"`
	// Desired offset to begin reading from. Value -1 has special handling, where
	// the read is performed from the current write head. All other positive
	// values specify a desired exact byte offset to read from. If the offset is
	// not available (eg, because it represents a portion of Journal which has
	// been permanently deleted), the broker will return the next available
	// offset. Callers should therefore always inspect the ReadResponse offset.
	Offset Offset `protobuf:"varint,3,opt,name=offset,proto3,casttype=Offset" json:"offset,omitempty"`
	// Whether the operation should block until content becomes available.
	// OFFSET_NOT_YET_AVAILABLE is returned if a non-blocking read has no ready
	// content.
	Block bool `protobuf:"varint,4,opt,name=block,proto3" json:"block,omitempty"`
	// If do_not_proxy is true, the broker will not proxy the read to another
	// broker, or open and proxy a remote Fragment on the client's behalf.
	DoNotProxy bool `protobuf:"varint,5,opt,name=do_not_proxy,json=doNotProxy,proto3" json:"do_not_proxy,omitempty"`
	// If metadata_only is true, the broker will respond with Journal and
	// Fragment metadata but not content.
	MetadataOnly bool `protobuf:"varint,6,opt,name=metadata_only,json=metadataOnly,proto3" json:"metadata_only,omitempty"`
	// Offset to read through. If zero, then the read end offset is unconstrained.
	EndOffset Offset `protobuf:"varint,7,opt,name=end_offset,json=endOffset,proto3,casttype=Offset" json:"end_offset,omitempty"`
}

ReadRequest is the unary request message of the broker Read RPC.

func (*ReadRequest) Descriptor

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

func (*ReadRequest) Equal added in v0.83.1

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

func (*ReadRequest) Marshal

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

func (*ReadRequest) MarshalTo

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

func (*ReadRequest) MarshalToSizedBuffer added in v0.86.1

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

func (*ReadRequest) ProtoMessage

func (*ReadRequest) ProtoMessage()

func (*ReadRequest) ProtoSize

func (m *ReadRequest) ProtoSize() (n int)

func (*ReadRequest) Reset

func (m *ReadRequest) Reset()

func (*ReadRequest) String

func (m *ReadRequest) String() string

func (*ReadRequest) Unmarshal

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

func (*ReadRequest) Validate

func (m *ReadRequest) Validate() error

Validate returns an error if the ReadRequest is not well-formed.

func (*ReadRequest) XXX_DiscardUnknown

func (m *ReadRequest) XXX_DiscardUnknown()

func (*ReadRequest) XXX_Marshal

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

func (*ReadRequest) XXX_Merge

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

func (*ReadRequest) XXX_Size

func (m *ReadRequest) XXX_Size() int

func (*ReadRequest) XXX_Unmarshal

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

type ReadResponse

type ReadResponse struct {
	// Status of the Read RPC.
	Status Status `protobuf:"varint,1,opt,name=status,proto3,enum=protocol.Status" json:"status,omitempty"`
	// Header of the response. Accompanies the first ReadResponse of the response
	// stream.
	Header *Header `protobuf:"bytes,2,opt,name=header,proto3" json:"header,omitempty"`
	// The effective offset of the read. See ReadRequest offset.
	Offset Offset `protobuf:"varint,3,opt,name=offset,proto3,casttype=Offset" json:"offset,omitempty"`
	// The offset to next be written, by the next append transaction served by
	// broker. In other words, the last offset through which content is
	// available to be read from the Journal. This is a metadata field and will
	// not be returned with a content response.
	WriteHead Offset `protobuf:"varint,4,opt,name=write_head,json=writeHead,proto3,casttype=Offset" json:"write_head,omitempty"`
	// Fragment to which the offset was mapped. This is a metadata field and will
	// not be returned with a content response.
	Fragment *Fragment `protobuf:"bytes,5,opt,name=fragment,proto3" json:"fragment,omitempty"`
	// If Fragment is remote, a URL from which it may be directly read.
	FragmentUrl string `protobuf:"bytes,6,opt,name=fragment_url,json=fragmentUrl,proto3" json:"fragment_url,omitempty"`
	// Content chunks of the read.
	Content []byte `protobuf:"bytes,7,opt,name=content,proto3" json:"content,omitempty"`
}

ReadResponse is the streamed response message of the broker Read RPC. Responses messages are of two types:

  • "Metadata" messages, which conveys the journal Fragment addressed by the request which is ready to be read.
  • "Chunk" messages, which carry associated journal Fragment content bytes.

A metadata message specifying a Fragment always precedes all "chunks" of the Fragment's content. Response streams may be very long lived, having many metadata and accompanying chunk messages. The reader may also block for long periods of time awaiting the next metadata message (eg, if the next offset hasn't yet committed). However once a metadata message is read, the reader is assured that its associated chunk messages are immediately forthcoming.

func (*ReadResponse) Descriptor

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

func (*ReadResponse) Equal added in v0.83.1

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

func (*ReadResponse) Marshal

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

func (*ReadResponse) MarshalTo

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

func (*ReadResponse) MarshalToSizedBuffer added in v0.86.1

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

func (*ReadResponse) ProtoMessage

func (*ReadResponse) ProtoMessage()

func (*ReadResponse) ProtoSize

func (m *ReadResponse) ProtoSize() (n int)

func (*ReadResponse) Reset

func (m *ReadResponse) Reset()

func (*ReadResponse) String

func (m *ReadResponse) String() string

func (*ReadResponse) Unmarshal

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

func (*ReadResponse) Validate

func (m *ReadResponse) Validate() error

Validate returns an error if the ReadResponse is not well-formed.

func (*ReadResponse) XXX_DiscardUnknown

func (m *ReadResponse) XXX_DiscardUnknown()

func (*ReadResponse) XXX_Marshal

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

func (*ReadResponse) XXX_Merge

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

func (*ReadResponse) XXX_Size

func (m *ReadResponse) XXX_Size() int

func (*ReadResponse) XXX_Unmarshal

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

type ReplicateRequest

type ReplicateRequest struct {
	// Header defines the primary broker, Route, and Etcd Revision under which
	// this Replicate stream is being established. Each replication peer
	// independently inspects and verifies the current Journal Route topology.
	Header *Header `protobuf:"bytes,1,opt,name=header,proto3" json:"header,omitempty"`
	// Proposed Fragment to commit, which is verified by each replica.
	Proposal *Fragment `protobuf:"bytes,3,opt,name=proposal,proto3" json:"proposal,omitempty"`
	// Registers proposed to apply, which are also verified by each replica.
	Registers *LabelSet `protobuf:"bytes,7,opt,name=registers,proto3" json:"registers,omitempty"`
	// Acknowledge requests that the peer send an acknowledging ReplicateResponse
	// on successful application of the ReplicateRequest.
	Acknowledge bool `protobuf:"varint,6,opt,name=acknowledge,proto3" json:"acknowledge,omitempty"`
	// Journal to be replicated to, which is also captured by |proposal|.
	// Deprecated.
	DeprecatedJournal Journal `` /* 129-byte string literal not displayed */
	// Content to be replicated.
	Content []byte `protobuf:"bytes,4,opt,name=content,proto3" json:"content,omitempty"`
	// Delta offset of |content| relative to current Fragment |end|.
	ContentDelta int64 `protobuf:"varint,5,opt,name=content_delta,json=contentDelta,proto3" json:"content_delta,omitempty"`
}

ReplicateRequest is the streamed request message of the broker's internal Replicate RPC. Each message is either a pending content chunk or a "proposal" to commit (or roll back) content chunks previously sent.

func (*ReplicateRequest) Descriptor

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

func (*ReplicateRequest) Equal added in v0.83.1

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

func (*ReplicateRequest) Marshal

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

func (*ReplicateRequest) MarshalTo

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

func (*ReplicateRequest) MarshalToSizedBuffer added in v0.86.1

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

func (*ReplicateRequest) ProtoMessage

func (*ReplicateRequest) ProtoMessage()

func (*ReplicateRequest) ProtoSize

func (m *ReplicateRequest) ProtoSize() (n int)

func (*ReplicateRequest) Reset

func (m *ReplicateRequest) Reset()

func (*ReplicateRequest) String

func (m *ReplicateRequest) String() string

func (*ReplicateRequest) Unmarshal

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

func (*ReplicateRequest) Validate

func (m *ReplicateRequest) Validate() error

Validate returns an error if the ReplicateRequest is not well-formed.

func (*ReplicateRequest) XXX_DiscardUnknown

func (m *ReplicateRequest) XXX_DiscardUnknown()

func (*ReplicateRequest) XXX_Marshal

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

func (*ReplicateRequest) XXX_Merge

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

func (*ReplicateRequest) XXX_Size

func (m *ReplicateRequest) XXX_Size() int

func (*ReplicateRequest) XXX_Unmarshal

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

type ReplicateResponse

type ReplicateResponse struct {
	// Status of the Replicate RPC.
	Status Status `protobuf:"varint,1,opt,name=status,proto3,enum=protocol.Status" json:"status,omitempty"`
	// Header of the response. Accompanies the first ReplicateResponse of the
	// response stream.
	Header *Header `protobuf:"bytes,2,opt,name=header,proto3" json:"header,omitempty"`
	// If status is PROPOSAL_MISMATCH, then |fragment| is the replica's current
	// journal Fragment, and either it or |registers| will differ from the
	// primary's proposal.
	Fragment *Fragment `protobuf:"bytes,3,opt,name=fragment,proto3" json:"fragment,omitempty"`
	// If status is PROPOSAL_MISMATCH, then |registers| are the replica's current
	// journal registers.
	Registers *LabelSet `protobuf:"bytes,4,opt,name=registers,proto3" json:"registers,omitempty"`
}

ReplicateResponse is the streamed response message of the broker's internal Replicate RPC. Each message is a 1:1 response to a previously read "proposal" ReplicateRequest with |acknowledge| set.

func (*ReplicateResponse) Descriptor

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

func (*ReplicateResponse) Equal added in v0.83.1

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

func (*ReplicateResponse) Marshal

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

func (*ReplicateResponse) MarshalTo

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

func (*ReplicateResponse) MarshalToSizedBuffer added in v0.86.1

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

func (*ReplicateResponse) ProtoMessage

func (*ReplicateResponse) ProtoMessage()

func (*ReplicateResponse) ProtoSize

func (m *ReplicateResponse) ProtoSize() (n int)

func (*ReplicateResponse) Reset

func (m *ReplicateResponse) Reset()

func (*ReplicateResponse) String

func (m *ReplicateResponse) String() string

func (*ReplicateResponse) Unmarshal

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

func (*ReplicateResponse) Validate

func (m *ReplicateResponse) Validate() error

Validate returns an error if the ReplicateResponse is not well-formed.

func (*ReplicateResponse) XXX_DiscardUnknown

func (m *ReplicateResponse) XXX_DiscardUnknown()

func (*ReplicateResponse) XXX_Marshal

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

func (*ReplicateResponse) XXX_Merge

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

func (*ReplicateResponse) XXX_Size

func (m *ReplicateResponse) XXX_Size() int

func (*ReplicateResponse) XXX_Unmarshal

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

type Route

type Route struct {
	// Members of the Route, ordered on ascending ProcessSpec.ID (zone, suffix).
	Members []ProcessSpec_ID `protobuf:"bytes,1,rep,name=members,proto3" json:"members"`
	// Index of the ProcessSpec serving as primary within |members|,
	// or -1 of no member is currently primary.
	Primary int32 `protobuf:"varint,2,opt,name=primary,proto3" json:"primary,omitempty"`
	// Endpoints of each Route member. If not empty, |endpoints| has the same
	// length and order as |members|, and captures the endpoint of each one.
	Endpoints []Endpoint `protobuf:"bytes,3,rep,name=endpoints,proto3,casttype=Endpoint" json:"endpoints,omitempty"`
}

Route captures the current topology of an item and the processes serving it.

func (Route) Copy

func (m Route) Copy() Route

Copy returns a deep copy of the Route.

func (*Route) Descriptor

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

func (*Route) Equal added in v0.83.1

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

func (Route) Equivalent

func (m Route) Equivalent(other *Route) bool

Equivalent returns true if the Routes have equivalent broker Names, Zones, and current Primary. It does not compare broker Endpoints.

func (*Route) Marshal

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

func (Route) MarshalString

func (m Route) MarshalString() string

MarshalString returns the marshaled encoding of the Route as a string.

func (*Route) MarshalTo

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

func (*Route) MarshalToSizedBuffer added in v0.86.1

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

func (*Route) ProtoMessage

func (*Route) ProtoMessage()

func (*Route) ProtoSize

func (m *Route) ProtoSize() (n int)

func (*Route) Reset

func (m *Route) Reset()

func (*Route) String

func (m *Route) String() string

func (*Route) Unmarshal

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

func (Route) Validate

func (m Route) Validate() error

Validate returns an error if the Route is not well-formed.

func (*Route) XXX_DiscardUnknown

func (m *Route) XXX_DiscardUnknown()

func (*Route) XXX_Marshal

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

func (*Route) XXX_Merge

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

func (*Route) XXX_Size

func (m *Route) XXX_Size() int

func (*Route) XXX_Unmarshal

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

type RoutedJournalClient

type RoutedJournalClient interface {
	JournalClient
	DispatchRouter
}

RoutedJournalClient composes a JournalClient and DispatchRouter.

func NewRoutedJournalClient

func NewRoutedJournalClient(jc JournalClient, dr DispatchRouter) RoutedJournalClient

NewRoutedJournalClient composes a JournalClient and DispatchRouter.

type SHA1Sum

type SHA1Sum struct {
	Part1 uint64 `protobuf:"fixed64,1,opt,name=part1,proto3" json:"part1,omitempty"`
	Part2 uint64 `protobuf:"fixed64,2,opt,name=part2,proto3" json:"part2,omitempty"`
	Part3 uint32 `protobuf:"fixed32,3,opt,name=part3,proto3" json:"part3,omitempty"`
}

SHA1Sum is a 160-bit SHA1 digest.

func SHA1SumFromDigest

func SHA1SumFromDigest(r []byte) SHA1Sum

SHA1SumFromDigest converts SHA1 sum in digest form into a SHA1Sum. |r| must have the length of a SHA1 digest (20 bytes), or it panics.

func SHA1SumOf

func SHA1SumOf(str string) SHA1Sum

SHA1SumOf SHA1 sums |str| and returns a SHA1Sum.

func (*SHA1Sum) Descriptor

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

func (*SHA1Sum) Equal added in v0.83.1

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

func (SHA1Sum) IsZero

func (m SHA1Sum) IsZero() bool

IsZero returns whether this SHA1Sum is zero-valued. As a special case, Fragments having no content are consistently mapped to the zero-valued SHA1Sum (rather than SHA1 of "", which is da39a3ee5e6b4b0d3255bfef95601890afd80709).

func (*SHA1Sum) Marshal

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

func (*SHA1Sum) MarshalTo

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

func (*SHA1Sum) MarshalToSizedBuffer added in v0.86.1

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

func (*SHA1Sum) ProtoMessage

func (*SHA1Sum) ProtoMessage()

func (*SHA1Sum) ProtoSize

func (m *SHA1Sum) ProtoSize() (n int)

func (*SHA1Sum) Reset

func (m *SHA1Sum) Reset()

func (*SHA1Sum) String

func (m *SHA1Sum) String() string

func (SHA1Sum) ToDigest

func (m SHA1Sum) ToDigest() (r [20]byte)

ToDigest converts the SHA1Sum to a flat, fixed-size array.

func (*SHA1Sum) Unmarshal

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

func (*SHA1Sum) XXX_DiscardUnknown

func (m *SHA1Sum) XXX_DiscardUnknown()

func (*SHA1Sum) XXX_Marshal

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

func (*SHA1Sum) XXX_Merge

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

func (*SHA1Sum) XXX_Size

func (m *SHA1Sum) XXX_Size() int

func (*SHA1Sum) XXX_Unmarshal

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

type Status

type Status int32

Status is a response status code, used universally across Gazette RPC APIs.

const (
	Status_OK Status = 0
	// The named journal does not exist.
	Status_JOURNAL_NOT_FOUND Status = 1
	// There is no current primary broker for the journal. This is a temporary
	// condition which should quickly resolve, assuming sufficient broker
	// capacity.
	Status_NO_JOURNAL_PRIMARY_BROKER Status = 2
	// The present broker is not the assigned primary broker for the journal.
	Status_NOT_JOURNAL_PRIMARY_BROKER Status = 3
	// The present broker is not an assigned broker for the journal.
	Status_NOT_JOURNAL_BROKER Status = 5
	// There are an insufficient number of assigned brokers for the journal
	// to meet its required replication.
	Status_INSUFFICIENT_JOURNAL_BROKERS Status = 4
	// The requested offset is not yet available. This indicates either that the
	// offset has not yet been written, or that the broker is not yet aware of a
	// written fragment covering the offset. Returned only by non-blocking reads.
	Status_OFFSET_NOT_YET_AVAILABLE Status = 6
	// The peer disagrees with the Route accompanying a ReplicateRequest.
	Status_WRONG_ROUTE Status = 7
	// The peer disagrees with the proposal accompanying a ReplicateRequest.
	Status_PROPOSAL_MISMATCH Status = 8
	// The Etcd transaction failed. Returned by Update RPC when an
	// expect_mod_revision of the UpdateRequest differs from the current
	// ModRevision of the JournalSpec within the store.
	Status_ETCD_TRANSACTION_FAILED Status = 9
	// A disallowed journal access was attempted (eg, a write where the
	// journal disables writes, or read where journals disable reads).
	Status_NOT_ALLOWED Status = 10
	// The Append is refused because its requested offset is not equal
	// to the furthest written offset of the journal.
	Status_WRONG_APPEND_OFFSET Status = 11
	// The Append is refused because the replication pipeline tracks a smaller
	// journal offset than that of the remote fragment index. This indicates
	// that journal replication consistency has been lost in the past, due to
	// too many broker or Etcd failures.
	Status_INDEX_HAS_GREATER_OFFSET Status = 12
	// The Append is refused because a registers selector was provided with the
	// request, but it was not matched by current register values of the journal.
	Status_REGISTER_MISMATCH Status = 13
)

func (Status) EnumDescriptor

func (Status) EnumDescriptor() ([]byte, []int)

func (Status) String

func (x Status) String() string

func (Status) Validate

func (x Status) Validate() error

type UnimplementedJournalServer added in v0.86.1

type UnimplementedJournalServer struct {
}

UnimplementedJournalServer can be embedded to have forward compatible implementations.

func (*UnimplementedJournalServer) Append added in v0.86.1

func (*UnimplementedJournalServer) Apply added in v0.86.1

func (*UnimplementedJournalServer) List added in v0.86.1

func (*UnimplementedJournalServer) ListFragments added in v0.86.1

func (*UnimplementedJournalServer) Read added in v0.86.1

func (*UnimplementedJournalServer) Replicate added in v0.86.1

type ValidationError

type ValidationError struct {
	Context []string
	Err     error
}

ValidationError is an error implementation which captures its validation context.

func (*ValidationError) Error

func (ve *ValidationError) Error() string

Error implements the error interface.

type Validator

type Validator interface {
	Validate() error
}

Validator is a type able to validate itself. Validate inspects the type for syntactic or semantic issues, and returns a descriptive error if any violations are encountered. It is recommended that Validate return instances of ValidationError where possible, which enables tracking nested contexts.

Directories

Path Synopsis
Package ext defines extensions to broker/protocol that depend on keyspace and allocator (which in turn depend on etcd).
Package ext defines extensions to broker/protocol that depend on keyspace and allocator (which in turn depend on etcd).

Jump to

Keyboard shortcuts

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