Documentation ¶
Index ¶
- Constants
- Variables
- func AppendCellFromRawData(dst *Column, rowData unsafe.Pointer, currentOffset int) int
- func Compare(row Row, colIdx int, ad *types.Datum) int
- func CopySelectedJoinRowsDirect(src *Chunk, selected []bool, dst *Chunk) (bool, error)
- func CopySelectedJoinRowsWithSameOuterRows(src *Chunk, innerColOffset, innerColLen, outerColOffset, outerColLen int, ...) (bool, error)
- func CopySelectedRows(dstCol *Column, srcCol *Column, selected []bool)
- func CopySelectedRowsWithRowIDFunc(dstCol *Column, srcCol *Column, selected []bool, start int, end int, ...)
- func EstimateTypeWidth(colType *types.FieldType) int
- func GetFixedLen(colType *types.FieldType) int
- func InitChunkAllocSize(setMaxFreeChunks, setMaxFreeColumns uint32)
- func NewAllocator() *allocator
- func NewIterator4RowContainer(c *RowContainer) *iterator4RowContainer
- func NewRowContainerReader(rc *RowContainer) *rowContainerReader
- type Allocator
- type Chunk
- func New(fields []*types.FieldType, capacity, maxChunkSize int) *Chunk
- func NewChunkFromPoolWithCapacity(fields []*types.FieldType, initCap int) *Chunk
- func NewChunkWithCapacity(fields []*types.FieldType, capacity int) *Chunk
- func NewEmptyChunk(fields []*types.FieldType) *Chunk
- func Renew(chk *Chunk, maxChunkSize int) *Chunk
- func (c *Chunk) Append(other *Chunk, begin, end int)
- func (c *Chunk) AppendBytes(colIdx int, b []byte)
- func (c *Chunk) AppendDatum(colIdx int, d *types.Datum)
- func (c *Chunk) AppendDuration(colIdx int, dur types.Duration)
- func (c *Chunk) AppendEnum(colIdx int, enum types.Enum)
- func (c *Chunk) AppendFloat32(colIdx int, f float32)
- func (c *Chunk) AppendFloat64(colIdx int, f float64)
- func (c *Chunk) AppendInt64(colIdx int, i int64)
- func (c *Chunk) AppendJSON(colIdx int, j types.BinaryJSON)
- func (c *Chunk) AppendMyDecimal(colIdx int, dec *types.MyDecimal)
- func (c *Chunk) AppendNull(colIdx int)
- func (c *Chunk) AppendPartialRow(colOff int, row Row)
- func (c *Chunk) AppendPartialRowByColIdxs(colOff int, row Row, colIdxs []int) (wide int)
- func (c *Chunk) AppendPartialRows(colOff int, rows []Row)
- func (c *Chunk) AppendRow(row Row)
- func (c *Chunk) AppendRowByColIdxs(row Row, colIdxs []int) (wide int)
- func (c *Chunk) AppendRows(rows []Row)
- func (c *Chunk) AppendSet(colIdx int, set types.Set)
- func (c *Chunk) AppendString(colIdx int, str string)
- func (c *Chunk) AppendTime(colIdx int, t types.Time)
- func (c *Chunk) AppendUint64(colIdx int, u uint64)
- func (c *Chunk) AppendVectorFloat32(colIdx int, v types.VectorFloat32)
- func (c *Chunk) Capacity() int
- func (c *Chunk) CloneEmpty(maxCapacity int) *Chunk
- func (c *Chunk) Column(colIdx int) *Column
- func (c *Chunk) CopyConstruct() *Chunk
- func (c *Chunk) CopyConstructSel() *Chunk
- func (c *Chunk) Destroy(initCap int, fields []*types.FieldType)
- func (c *Chunk) GetNumVirtualRows() int
- func (c *Chunk) GetRow(idx int) Row
- func (c *Chunk) GrowAndReset(maxChunkSize int)
- func (c *Chunk) IsFull() bool
- func (c *Chunk) IsInCompleteChunk() bool
- func (c *Chunk) LowerBound(colIdx int, d *types.Datum) (index int, match bool)
- func (c *Chunk) MakeRef(srcColIdx, dstColIdx int)
- func (c *Chunk) MakeRefTo(dstColIdx int, src *Chunk, srcColIdx int) error
- func (c *Chunk) MemoryUsage() (sum int64)
- func (c *Chunk) NumCols() int
- func (c *Chunk) NumRows() int
- func (c *Chunk) Prune(usedColIdxs []int) *Chunk
- func (c *Chunk) Reconstruct()
- func (c *Chunk) RequiredRows() int
- func (c *Chunk) Reset()
- func (c *Chunk) Sel() []int
- func (c *Chunk) SetCol(colIdx int, col *Column) *Column
- func (c *Chunk) SetInCompleteChunk(isInCompleteChunk bool)
- func (c *Chunk) SetNumVirtualRows(numVirtualRows int)
- func (c *Chunk) SetRequiredRows(requiredRows, maxChunkSize int) *Chunk
- func (c *Chunk) SetSel(sel []int)
- func (c *Chunk) SwapColumns(other *Chunk)
- func (c *Chunk) ToString(ft []*types.FieldType) string
- func (c *Chunk) TruncateTo(numRows int)
- func (c *Chunk) UpperBound(colIdx int, d *types.Datum) int
- type Codec
- type Column
- func (c *Column) AppendBytes(b []byte)
- func (c *Column) AppendCellNTimes(src *Column, pos, times int)
- func (c *Column) AppendDuration(dur types.Duration)
- func (c *Column) AppendEnum(enum types.Enum)
- func (c *Column) AppendFloat32(f float32)
- func (c *Column) AppendFloat64(f float64)
- func (c *Column) AppendInt64(i int64)
- func (c *Column) AppendJSON(j types.BinaryJSON)
- func (c *Column) AppendMyDecimal(dec *types.MyDecimal)
- func (c *Column) AppendNNulls(n int)
- func (c *Column) AppendNull()
- func (c *Column) AppendNullBitmap(notNull bool)
- func (c *Column) AppendSet(set types.Set)
- func (c *Column) AppendString(str string)
- func (c *Column) AppendTime(t types.Time)
- func (c *Column) AppendUint64(u uint64)
- func (c *Column) AppendVectorFloat32(v types.VectorFloat32)
- func (c *Column) ContainsVeryLargeElement() bool
- func (c *Column) CopyConstruct(dst *Column) *Column
- func (c *Column) CopyReconstruct(sel []int, dst *Column) *Column
- func (c *Column) Decimals() []types.MyDecimal
- func (c *Column) DestroyDataForTest()
- func (c *Column) Float32s() []float32
- func (c *Column) Float64s() []float64
- func (c *Column) GetBytes(rowID int) []byte
- func (c *Column) GetDecimal(rowID int) *types.MyDecimal
- func (c *Column) GetDuration(rowID int, fillFsp int) types.Duration
- func (c *Column) GetEnum(rowID int) types.Enum
- func (c *Column) GetFloat32(rowID int) float32
- func (c *Column) GetFloat64(rowID int) float64
- func (c *Column) GetInt64(rowID int) int64
- func (c *Column) GetJSON(rowID int) types.BinaryJSON
- func (c *Column) GetRaw(rowID int) []byte
- func (c *Column) GetSet(rowID int) types.Set
- func (c *Column) GetString(rowID int) string
- func (c *Column) GetTime(rowID int) types.Time
- func (c *Column) GetUint64(rowID int) uint64
- func (c *Column) GetVectorFloat32(rowID int) types.VectorFloat32
- func (c *Column) GoDurations() []time.Duration
- func (c *Column) Int64s() []int64
- func (c *Column) IsNull(rowIdx int) bool
- func (c *Column) MergeNulls(cols ...*Column)
- func (c *Column) ReserveBytes(n int)
- func (c *Column) ReserveEnum(n int)
- func (c *Column) ReserveJSON(n int)
- func (c *Column) ReserveSet(n int)
- func (c *Column) ReserveString(n int)
- func (c *Column) ReserveVectorFloat32(n int)
- func (c *Column) Reset(eType types.EvalType)
- func (c *Column) ResizeDecimal(n int, isNull bool)
- func (c *Column) ResizeFloat32(n int, isNull bool)
- func (c *Column) ResizeFloat64(n int, isNull bool)
- func (c *Column) ResizeGoDuration(n int, isNull bool)
- func (c *Column) ResizeInt64(n int, isNull bool)
- func (c *Column) ResizeTime(n int, isNull bool)
- func (c *Column) ResizeUint64(n int, isNull bool)
- func (c *Column) Rows() int
- func (c *Column) SetNull(rowIdx int, isNull bool)
- func (c *Column) SetNulls(begin, end int, isNull bool)
- func (c *Column) SetRaw(rowID int, bs []byte)
- func (c *Column) Times() []types.Time
- func (c *Column) Uint64s() []uint64
- type ColumnAllocator
- type ColumnSwapHelper
- type CompareFunc
- type DataInDiskByChunks
- func (d *DataInDiskByChunks) Add(chk *Chunk) (err error)
- func (d *DataInDiskByChunks) Close()
- func (d *DataInDiskByChunks) GetChunk(chkIdx int) (*Chunk, error)
- func (d *DataInDiskByChunks) GetDiskTracker() *disk.Tracker
- func (d *DataInDiskByChunks) NumChunks() int
- func (d *DataInDiskByChunks) NumRows() int64
- type DataInDiskByRows
- func (l *DataInDiskByRows) Add(chk *Chunk) (err error)
- func (l *DataInDiskByRows) Close() error
- func (l *DataInDiskByRows) GetChunk(chkIdx int) (*Chunk, error)
- func (l *DataInDiskByRows) GetDiskTracker() *disk.Tracker
- func (l *DataInDiskByRows) GetRow(ptr RowPtr) (row Row, err error)
- func (l *DataInDiskByRows) GetRowAndAppendToChunk(ptr RowPtr, chk *Chunk) (row Row, _ *Chunk, err error)
- func (l *DataInDiskByRows) Len() int
- func (l *DataInDiskByRows) NumChunks() int
- func (l *DataInDiskByRows) NumRowsOfChunk(chkID int) int
- type Decoder
- type DefaultColumnAllocator
- type Iterator
- type Iterator4Chunk
- func (it *Iterator4Chunk) Begin() Row
- func (it *Iterator4Chunk) Current() Row
- func (*Iterator4Chunk) End() Row
- func (*Iterator4Chunk) Error() error
- func (it *Iterator4Chunk) GetChunk() *Chunk
- func (it *Iterator4Chunk) Len() int
- func (it *Iterator4Chunk) Next() Row
- func (it *Iterator4Chunk) ReachEnd()
- func (it *Iterator4Chunk) ResetChunk(chk *Chunk)
- type Iterator4Slice
- func (it *Iterator4Slice) Begin() Row
- func (it *Iterator4Slice) Current() Row
- func (*Iterator4Slice) End() Row
- func (*Iterator4Slice) Error() error
- func (it *Iterator4Slice) Len() int
- func (it *Iterator4Slice) Next() Row
- func (it *Iterator4Slice) ReachEnd()
- func (it *Iterator4Slice) Reset(rows []Row)
- type List
- func (l *List) Add(chk *Chunk)
- func (l *List) AppendRow(row Row) RowPtr
- func (l *List) Clear()
- func (l *List) FieldTypes() []*types.FieldType
- func (l *List) GetChunk(chkIdx int) *Chunk
- func (l *List) GetMemTracker() *memory.Tracker
- func (l *List) GetRow(ptr RowPtr) Row
- func (l *List) Len() int
- func (l *List) NumChunks() int
- func (l *List) NumRowsOfChunk(chkID int) int
- func (l *List) Reset()
- func (l *List) Walk(walkFunc ListWalkFunc) error
- type ListWalkFunc
- type MutRow
- func (mr MutRow) Clone() MutRow
- func (mr MutRow) Len() int
- func (mr MutRow) SetDatum(colIdx int, d types.Datum)
- func (mr MutRow) SetDatums(datums ...types.Datum)
- func (mr MutRow) SetRow(row Row)
- func (mr MutRow) SetValue(colIdx int, val any)
- func (mr MutRow) SetValues(vals ...any)
- func (mr MutRow) ShallowCopyPartialRow(colIdx int, row Row)
- func (mr MutRow) ToRow() Row
- type Pool
- type ReaderWithCache
- type Row
- func (r Row) Chunk() *Chunk
- func (r Row) CopyConstruct() Row
- func (r Row) DatumWithBuffer(colIdx int, tp *types.FieldType, d *types.Datum)
- func (r Row) GetBytes(colIdx int) []byte
- func (r Row) GetDatum(colIdx int, tp *types.FieldType) types.Datum
- func (r Row) GetDatumRow(fields []*types.FieldType) []types.Datum
- func (r Row) GetDatumRowWithBuffer(fields []*types.FieldType, datumRow []types.Datum) []types.Datum
- func (r Row) GetDuration(colIdx int, fillFsp int) types.Duration
- func (r Row) GetEnum(colIdx int) types.Enum
- func (r Row) GetFloat32(colIdx int) float32
- func (r Row) GetFloat64(colIdx int) float64
- func (r Row) GetInt64(colIdx int) int64
- func (r Row) GetJSON(colIdx int) types.BinaryJSON
- func (r Row) GetMyDecimal(colIdx int) *types.MyDecimal
- func (r Row) GetRaw(colIdx int) []byte
- func (r Row) GetSet(colIdx int) types.Set
- func (r Row) GetString(colIdx int) string
- func (r Row) GetTime(colIdx int) types.Time
- func (r Row) GetUint64(colIdx int) uint64
- func (r Row) GetVectorFloat32(colIdx int) types.VectorFloat32
- func (r Row) Idx() int
- func (r Row) IsEmpty() bool
- func (r Row) IsNull(colIdx int) bool
- func (r Row) Len() int
- func (r Row) ToString(ft []*types.FieldType) string
- type RowContainer
- func (c *RowContainer) ActionSpill() *SpillDiskAction
- func (c *RowContainer) ActionSpillForTest() *SpillDiskAction
- func (c *RowContainer) Add(chk *Chunk) (err error)
- func (c *RowContainer) AllocChunk() (chk *Chunk)
- func (c *RowContainer) AlreadySpilledSafeForTest() bool
- func (c *RowContainer) Close() (err error)
- func (c *RowContainer) GetChunk(chkIdx int) (*Chunk, error)
- func (c *RowContainer) GetDiskTracker() *disk.Tracker
- func (c *RowContainer) GetMemTracker() *memory.Tracker
- func (c *RowContainer) GetRow(ptr RowPtr) (row Row, err error)
- func (c *RowContainer) GetRowAndAlwaysAppendToChunk(ptr RowPtr, chk *Chunk) (row Row, _ *Chunk, err error)
- func (c *RowContainer) GetRowAndAppendToChunkIfInDisk(ptr RowPtr, chk *Chunk) (row Row, _ *Chunk, err error)
- func (c *RowContainer) NumChunks() int
- func (c *RowContainer) NumRow() int
- func (c *RowContainer) NumRowsOfChunk(chkID int) int
- func (c *RowContainer) Reset() error
- func (c *RowContainer) ShallowCopyWithNewMutex() *RowContainer
- func (c *RowContainer) SpillToDisk()
- type RowContainerReader
- type RowPtr
- type SortAndSpillDiskAction
- type SortedRowContainer
- func (c *SortedRowContainer) ActionSpill() *SortAndSpillDiskAction
- func (c *SortedRowContainer) ActionSpillForTest() *SortAndSpillDiskAction
- func (c *SortedRowContainer) Add(chk *Chunk) (err error)
- func (c *SortedRowContainer) Close() error
- func (c *SortedRowContainer) GetMemTracker() *memory.Tracker
- func (c *SortedRowContainer) GetSortedRow(idx int) (Row, error)
- func (c *SortedRowContainer) GetSortedRowAndAlwaysAppendToChunk(idx int, chk *Chunk) (Row, *Chunk, error)
- func (c *SortedRowContainer) Sort() (ret error)
- func (c *SortedRowContainer) SpillToDisk()
- type SpillDiskAction
Constants ¶
const ( InitialCapacity = 32 ZeroCapacity = 0 )
Capacity constants.
const RowPtrSize = int(unsafe.Sizeof(RowPtr{}))
RowPtrSize shows the size of RowPtr
const RowSize = int64(unsafe.Sizeof(Row{}))
RowSize is the size of `Row{}`
const SignalCheckpointForSort uint = 10240
SignalCheckpointForSort indicates the times of row comparation that a signal detection will be triggered.
const VarElemLen = -1
VarElemLen indicates this Column is a variable length Column.
Variables ¶
var ErrCannotAddBecauseSorted = errors.New("can not add because sorted")
ErrCannotAddBecauseSorted indicate that the SortPartition is sorted and prohibit inserting data.
var MaxCachedLen = 16 * 1024
MaxCachedLen Maximum cacheable length
Functions ¶
func AppendCellFromRawData ¶
AppendCellFromRawData appends the cell from raw data
func Compare ¶
Compare compares the value with ad. We assume that the collation information of the column is the same with the datum.
func CopySelectedJoinRowsDirect ¶
CopySelectedJoinRowsDirect directly copies the selected joined rows from the source Chunk to the destination Chunk. Return true if at least one joined row was selected.
func CopySelectedJoinRowsWithSameOuterRows ¶
func CopySelectedJoinRowsWithSameOuterRows(src *Chunk, innerColOffset, innerColLen, outerColOffset, outerColLen int, selected []bool, dst *Chunk) (bool, error)
CopySelectedJoinRowsWithSameOuterRows copies the selected joined rows from the source Chunk to the destination Chunk. Return true if at least one joined row was selected.
NOTE: All the outer rows in the source Chunk should be the same.
func CopySelectedRows ¶
CopySelectedRows copies the selected rows in srcCol to dstCol
func CopySelectedRowsWithRowIDFunc ¶
func CopySelectedRowsWithRowIDFunc(dstCol *Column, srcCol *Column, selected []bool, start int, end int, rowIDFunc func(int) int)
CopySelectedRowsWithRowIDFunc copies the selected rows in srcCol to dstCol
func EstimateTypeWidth ¶
EstimateTypeWidth estimates the average width of values of the type. This is used by the planner, which doesn't require absolutely correct results; it's OK (and expected) to guess if we don't know for sure.
mostly study from https://github.com/postgres/postgres/blob/REL_12_STABLE/src/backend/utils/cache/lsyscache.c#L2356
func GetFixedLen ¶
GetFixedLen get the memory size of a fixed-length type. if colType is not fixed-length, it returns VarElemLen, aka -1.
func InitChunkAllocSize ¶
func InitChunkAllocSize(setMaxFreeChunks, setMaxFreeColumns uint32)
InitChunkAllocSize init the maximum cache size
func NewIterator4RowContainer ¶
func NewIterator4RowContainer(c *RowContainer) *iterator4RowContainer
NewIterator4RowContainer create a new iterator for RowContainer
func NewRowContainerReader ¶
func NewRowContainerReader(rc *RowContainer) *rowContainerReader
NewRowContainerReader creates a forward only iterator for row container
Types ¶
type Allocator ¶
type Allocator interface { Alloc(fields []*types.FieldType, capacity, maxChunkSize int) *Chunk CheckReuseAllocSize() bool Reset() }
Allocator is an interface defined to reduce object allocation. The typical usage is to call Reset() to recycle objects into a pool, and Alloc() allocates from the pool.
func NewEmptyAllocator ¶
func NewEmptyAllocator() Allocator
NewEmptyAllocator creates an empty pool, which will always call `chunk.New` to create a new chunk
func NewReuseHookAllocator ¶
NewReuseHookAllocator creates an allocator, which will call the function `f` when the first reused chunk is allocated.
func NewSyncAllocator ¶
NewSyncAllocator creates the synchronized version of the `alloc`
type Chunk ¶
type Chunk struct {
// contains filtered or unexported fields
}
Chunk stores multiple rows of data in columns. Columns are in Apache Arrow format. See https://arrow.apache.org/docs/format/Columnar.html#physical-memory-layout. Apache Arrow is not used directly because we want to access MySQL types without decoding.
Values are appended in compact format and can be directly accessed without decoding. When the chunk is done processing, we can reuse the allocated memory by resetting it.
All Chunk's API should not do the validation work, and the user should ensure it is used correctly.
func New ¶
New creates a new chunk.
cap: the limit for the max number of rows. maxChunkSize: the max limit for the number of rows.
func NewChunkFromPoolWithCapacity ¶
NewChunkFromPoolWithCapacity creates a new chunk with field types and capacity from the pool.
func NewChunkWithCapacity ¶
NewChunkWithCapacity creates a new chunk with field types and capacity.
func NewEmptyChunk ¶
NewEmptyChunk creates an empty chunk
func Renew ¶
Renew creates a new Chunk based on an existing Chunk. The newly created Chunk has the same data schema with the old Chunk. The capacity of the new Chunk might be doubled based on the capacity of the old Chunk and the maxChunkSize.
chk: old chunk(often used in previous call). maxChunkSize: the limit for the max number of rows.
func (*Chunk) AppendBytes ¶
AppendBytes appends a bytes value to the chunk.
func (*Chunk) AppendDatum ¶
AppendDatum appends a datum into the chunk.
func (*Chunk) AppendDuration ¶
AppendDuration appends a Duration value to the chunk. Fsp is ignored.
func (*Chunk) AppendEnum ¶
AppendEnum appends an Enum value to the chunk.
func (*Chunk) AppendFloat32 ¶
AppendFloat32 appends a float32 value to the chunk.
func (*Chunk) AppendFloat64 ¶
AppendFloat64 appends a float64 value to the chunk.
func (*Chunk) AppendInt64 ¶
AppendInt64 appends a int64 value to the chunk.
func (*Chunk) AppendJSON ¶
func (c *Chunk) AppendJSON(colIdx int, j types.BinaryJSON)
AppendJSON appends a JSON value to the chunk.
func (*Chunk) AppendMyDecimal ¶
AppendMyDecimal appends a MyDecimal value to the chunk.
func (*Chunk) AppendNull ¶
AppendNull appends a null value to the chunk.
func (*Chunk) AppendPartialRow ¶
AppendPartialRow appends a row to the chunk, starting from colOff.
func (*Chunk) AppendPartialRowByColIdxs ¶
AppendPartialRowByColIdxs appends a row to the chunk starting from colOff, using the row's columns specified by colIdxs. 1. every columns are used if colIdxs is nil. 2. no columns are used if colIdxs is not nil but the size of colIdxs is 0.
func (*Chunk) AppendPartialRows ¶
AppendPartialRows appends multiple rows to the chunk.
func (*Chunk) AppendRowByColIdxs ¶
AppendRowByColIdxs appends a row to the chunk, using the row's columns specified by colIdxs. 1. every columns are used if colIdxs is nil. 2. no columns are used if colIdxs is not nil but the size of colIdxs is 0.
func (*Chunk) AppendRows ¶
AppendRows appends multiple rows to the chunk.
func (*Chunk) AppendString ¶
AppendString appends a string value to the chunk.
func (*Chunk) AppendTime ¶
AppendTime appends a Time value to the chunk.
func (*Chunk) AppendUint64 ¶
AppendUint64 appends a uint64 value to the chunk.
func (*Chunk) AppendVectorFloat32 ¶
func (c *Chunk) AppendVectorFloat32(colIdx int, v types.VectorFloat32)
AppendVectorFloat32 appends a VectorFloat32 value to the chunk.
func (*Chunk) CloneEmpty ¶
CloneEmpty returns an empty chunk that has the same schema with current chunk
func (*Chunk) CopyConstruct ¶
CopyConstruct creates a new chunk and copies this chunk's data into it.
func (*Chunk) CopyConstructSel ¶
CopyConstructSel is just like CopyConstruct, but ignore the rows that was not selected.
func (*Chunk) GetNumVirtualRows ¶
GetNumVirtualRows return c.numVirtualRows, used only in test
func (*Chunk) GrowAndReset ¶
GrowAndReset resets the Chunk and doubles the capacity of the Chunk. The doubled capacity should not be larger than maxChunkSize.
func (*Chunk) IsInCompleteChunk ¶
IsInCompleteChunk returns true if this chunk is inCompleteChunk, used only in test
func (*Chunk) LowerBound ¶
LowerBound searches on the non-decreasing Column colIdx, returns the smallest index i such that the value at row i is not less than `d`.
func (*Chunk) MakeRefTo ¶
MakeRefTo copies columns `src.columns[srcColIdx]` to `c.columns[dstColIdx]`.
func (*Chunk) MemoryUsage ¶
MemoryUsage returns the total memory usage of a Chunk in bytes. We ignore the size of Column.length and Column.nullCount since they have little effect of the total memory usage.
func (*Chunk) Prune ¶
Prune creates a new Chunk according to `c` and prunes the columns whose index is not in `usedColIdxs`
func (*Chunk) Reconstruct ¶
func (c *Chunk) Reconstruct()
Reconstruct removes all filtered rows in this Chunk.
func (*Chunk) RequiredRows ¶
RequiredRows returns how many rows is considered full.
func (*Chunk) Reset ¶
func (c *Chunk) Reset()
Reset resets the chunk, so the memory it allocated can be reused. Make sure all the data in the chunk is not used anymore before you reuse this chunk.
func (*Chunk) SetInCompleteChunk ¶
SetInCompleteChunk will set c.inCompleteChunk, used in join
func (*Chunk) SetNumVirtualRows ¶
SetNumVirtualRows sets the virtual row number for a Chunk. It should only be used when there exists no Column in the Chunk.
func (*Chunk) SetRequiredRows ¶
SetRequiredRows sets the number of required rows.
func (*Chunk) SwapColumns ¶
SwapColumns swaps columns with another Chunk.
func (*Chunk) TruncateTo ¶
TruncateTo truncates rows from tail to head in a Chunk to "numRows" rows.
type Codec ¶
type Codec struct {
// contains filtered or unexported fields
}
Codec is used to: 1. encode a Chunk to a byte slice. 2. decode a Chunk from a byte slice.
func (*Codec) DecodeToChunk ¶
DecodeToChunk decodes a Chunk from a byte slice, return the remained unused bytes.
type Column ¶
type Column struct {
// contains filtered or unexported fields
}
Column stores one column of data in Apache Arrow format. See https://arrow.apache.org/docs/format/Columnar.html#format-columnar
func NewEmptyColumn ¶
NewEmptyColumn creates a new column with nothing.
func (*Column) AppendBytes ¶
AppendBytes appends a byte slice into this Column.
func (*Column) AppendCellNTimes ¶
AppendCellNTimes append the pos-th Cell in source column to target column N times
func (*Column) AppendDuration ¶
AppendDuration appends a duration value into this Column. Fsp is ignored.
func (*Column) AppendEnum ¶
AppendEnum appends a Enum value into this Column.
func (*Column) AppendFloat32 ¶
AppendFloat32 appends a float32 value into this Column.
func (*Column) AppendFloat64 ¶
AppendFloat64 appends a float64 value into this Column.
func (*Column) AppendInt64 ¶
AppendInt64 appends an int64 value into this Column.
func (*Column) AppendJSON ¶
func (c *Column) AppendJSON(j types.BinaryJSON)
AppendJSON appends a BinaryJSON value into this Column.
func (*Column) AppendMyDecimal ¶
AppendMyDecimal appends a MyDecimal value into this Column.
func (*Column) AppendNNulls ¶
AppendNNulls append n nulls to the column
func (*Column) AppendNull ¶
func (c *Column) AppendNull()
AppendNull appends a null value into this Column.
func (*Column) AppendNullBitmap ¶
AppendNullBitmap append a null/notnull value to the column's null map
func (*Column) AppendString ¶
AppendString appends a string value into this Column.
func (*Column) AppendTime ¶
AppendTime appends a time value into this Column.
func (*Column) AppendUint64 ¶
AppendUint64 appends a uint64 value into this Column.
func (*Column) AppendVectorFloat32 ¶
func (c *Column) AppendVectorFloat32(v types.VectorFloat32)
AppendVectorFloat32 appends a VectorFloat32 value into this Column.
func (*Column) ContainsVeryLargeElement ¶
ContainsVeryLargeElement checks if the column contains element whose length is greater than math.MaxUint32.
func (*Column) CopyConstruct ¶
CopyConstruct copies this Column to dst. If dst is nil, it creates a new Column and returns it.
func (*Column) CopyReconstruct ¶
CopyReconstruct copies this Column to dst and removes unselected rows. If dst is nil, it creates a new Column and returns it.
func (*Column) DestroyDataForTest ¶
func (c *Column) DestroyDataForTest()
DestroyDataForTest destroies data in the column so that we can test if data in column are deeply copied.
func (*Column) GetDecimal ¶
GetDecimal returns the decimal in the specific row.
func (*Column) GetDuration ¶
GetDuration returns the Duration in the specific row.
func (*Column) GetFloat32 ¶
GetFloat32 returns the float32 in the specific row.
func (*Column) GetFloat64 ¶
GetFloat64 returns the float64 in the specific row.
func (*Column) GetJSON ¶
func (c *Column) GetJSON(rowID int) types.BinaryJSON
GetJSON returns the JSON in the specific row.
func (*Column) GetVectorFloat32 ¶
func (c *Column) GetVectorFloat32(rowID int) types.VectorFloat32
GetVectorFloat32 returns the VectorFloat32 in the specific row.
func (*Column) GoDurations ¶
GoDurations returns a Golang time.Duration slice stored in this Column. Different from the Row.GetDuration method, the argument Fsp is ignored, so the user should handle it outside.
func (*Column) MergeNulls ¶
MergeNulls merges these columns' null bitmaps. For a row, if any column of it is null, the result is null. It works like: if col1.IsNull || col2.IsNull || col3.IsNull. The caller should ensure that all these columns have the same length, and data stored in the result column is fixed-length type.
func (*Column) ReserveBytes ¶
ReserveBytes changes the column capacity to store n bytes elements and set the length to zero.
func (*Column) ReserveEnum ¶
ReserveEnum changes the column capacity to store n enum elements and set the length to zero.
func (*Column) ReserveJSON ¶
ReserveJSON changes the column capacity to store n JSON elements and set the length to zero.
func (*Column) ReserveSet ¶
ReserveSet changes the column capacity to store n set elements and set the length to zero.
func (*Column) ReserveString ¶
ReserveString changes the column capacity to store n string elements and set the length to zero.
func (*Column) ReserveVectorFloat32 ¶
ReserveVectorFloat32 changes the column capacity to store n vectorFloat32 elements and set the length to zero.
func (*Column) Reset ¶
Reset resets this Column according to the EvalType. Different from reset, Reset will reset the elemBuf.
func (*Column) ResizeDecimal ¶
ResizeDecimal resizes the column so that it contains n decimal elements.
func (*Column) ResizeFloat32 ¶
ResizeFloat32 resizes the column so that it contains n float32 elements.
func (*Column) ResizeFloat64 ¶
ResizeFloat64 resizes the column so that it contains n float64 elements.
func (*Column) ResizeGoDuration ¶
ResizeGoDuration resizes the column so that it contains n duration elements.
func (*Column) ResizeInt64 ¶
ResizeInt64 resizes the column so that it contains n int64 elements.
func (*Column) ResizeTime ¶
ResizeTime resizes the column so that it contains n Time elements.
func (*Column) ResizeUint64 ¶
ResizeUint64 resizes the column so that it contains n uint64 elements.
func (*Column) SetRaw ¶
SetRaw sets the raw bytes for the rowIdx-th element. NOTE: Two conditions must be satisfied before calling this function: 1. The column should be stored with variable-length elements. 2. The length of the new element should be exactly the same as the old one.
type ColumnAllocator ¶
ColumnAllocator defines an allocator for Column.
type ColumnSwapHelper ¶
type ColumnSwapHelper struct { // InputIdxToOutputIdxes maps the input column index to the output column indexes. InputIdxToOutputIdxes map[int][]int // contains filtered or unexported fields }
ColumnSwapHelper is used to help swap columns in a chunk.
func NewColumnSwapHelper ¶
func NewColumnSwapHelper(usedColumnIndex []int) *ColumnSwapHelper
NewColumnSwapHelper creates a new ColumnSwapHelper.
func (*ColumnSwapHelper) SwapColumns ¶
func (helper *ColumnSwapHelper) SwapColumns(input, output *Chunk) error
SwapColumns evaluates "Column" expressions. it will change the content of the input Chunk.
type CompareFunc ¶
CompareFunc is a function to compare the two values in Row, the two columns must have the same type.
func GetCompareFunc ¶
func GetCompareFunc(tp *types.FieldType) CompareFunc
GetCompareFunc gets a compare function for the field type.
type DataInDiskByChunks ¶
type DataInDiskByChunks struct {
// contains filtered or unexported fields
}
DataInDiskByChunks represents some data stored in temporary disk. They can only be restored by chunks.
func NewDataInDiskByChunks ¶
func NewDataInDiskByChunks(fieldTypes []*types.FieldType) *DataInDiskByChunks
NewDataInDiskByChunks creates a new DataInDiskByChunks with field types.
func (*DataInDiskByChunks) Add ¶
func (d *DataInDiskByChunks) Add(chk *Chunk) (err error)
Add adds a chunk to the DataInDiskByChunks. Caller must make sure the input chk has the same field types. Warning: Do not concurrently call this function.
func (*DataInDiskByChunks) Close ¶
func (d *DataInDiskByChunks) Close()
Close releases the disk resource.
func (*DataInDiskByChunks) GetChunk ¶
func (d *DataInDiskByChunks) GetChunk(chkIdx int) (*Chunk, error)
GetChunk gets a Chunk from the DataInDiskByChunks by chkIdx.
func (*DataInDiskByChunks) GetDiskTracker ¶
func (d *DataInDiskByChunks) GetDiskTracker() *disk.Tracker
GetDiskTracker returns the memory tracker of this List.
func (*DataInDiskByChunks) NumChunks ¶
func (d *DataInDiskByChunks) NumChunks() int
NumChunks returns total spilled chunk number
func (*DataInDiskByChunks) NumRows ¶
func (d *DataInDiskByChunks) NumRows() int64
NumRows returns total spilled row number
type DataInDiskByRows ¶
type DataInDiskByRows struct {
// contains filtered or unexported fields
}
DataInDiskByRows represents some data stored in temporary disk. These data are stored in row format, so they can only be restored by rows.
func NewDataInDiskByRows ¶
func NewDataInDiskByRows(fieldTypes []*types.FieldType) *DataInDiskByRows
NewDataInDiskByRows creates a new DataInDiskByRows with field types.
func (*DataInDiskByRows) Add ¶
func (l *DataInDiskByRows) Add(chk *Chunk) (err error)
Add adds a chunk to the DataInDiskByRows. Caller must make sure the input chk is not empty and not used any more and has the same field types. Warning: Do not use Add concurrently.
func (*DataInDiskByRows) Close ¶
func (l *DataInDiskByRows) Close() error
Close releases the disk resource.
func (*DataInDiskByRows) GetChunk ¶
func (l *DataInDiskByRows) GetChunk(chkIdx int) (*Chunk, error)
GetChunk gets a Chunk from the DataInDiskByRows by chkIdx.
func (*DataInDiskByRows) GetDiskTracker ¶
func (l *DataInDiskByRows) GetDiskTracker() *disk.Tracker
GetDiskTracker returns the memory tracker of this List.
func (*DataInDiskByRows) GetRow ¶
func (l *DataInDiskByRows) GetRow(ptr RowPtr) (row Row, err error)
GetRow gets a Row from the DataInDiskByRows by RowPtr.
func (*DataInDiskByRows) GetRowAndAppendToChunk ¶
func (l *DataInDiskByRows) GetRowAndAppendToChunk(ptr RowPtr, chk *Chunk) (row Row, _ *Chunk, err error)
GetRowAndAppendToChunk gets a Row from the DataInDiskByRows by RowPtr. Return the Row and the Ref Chunk.
func (*DataInDiskByRows) Len ¶
func (l *DataInDiskByRows) Len() int
Len returns the number of rows in DataInDiskByRows
func (*DataInDiskByRows) NumChunks ¶
func (l *DataInDiskByRows) NumChunks() int
NumChunks returns the number of chunks in the DataInDiskByRows.
func (*DataInDiskByRows) NumRowsOfChunk ¶
func (l *DataInDiskByRows) NumRowsOfChunk(chkID int) int
NumRowsOfChunk returns the number of rows of a chunk in the DataInDiskByRows.
type Decoder ¶
type Decoder struct {
// contains filtered or unexported fields
}
Decoder decodes the data returned from the coprocessor and stores the result in Chunk. How Decoder works:
- Initialization phase: Decode a whole input byte slice to Decoder.intermChk(intermediate chunk) using Codec.Decode. intermChk is introduced to simplify the implementation of decode phase. This phase uses pointer operations with less CPU and memory cost.
- Decode phase: 2.1 Set the number of rows to be decoded to a value that is a multiple of 8 and greater than `chk.RequiredRows() - chk.NumRows()`. This reduces the overhead of copying the srcCol.nullBitMap into destCol.nullBitMap. 2.2 Append srcCol.offsets to destCol.offsets when the elements is of var-length type. And further adjust the offsets according to descCol.offsets[destCol.length]-srcCol.offsets[0]. 2.3 Append srcCol.nullBitMap to destCol.nullBitMap.
- Go to step 1 when the input byte slice is consumed.
func NewDecoder ¶
NewDecoder creates a new Decoder object for decode a Chunk.
func (*Decoder) Decode ¶
Decode decodes multiple rows of Decoder.intermChk and stores the result in chk.
func (*Decoder) IsFinished ¶
IsFinished indicates whether Decoder.intermChk has been dried up.
func (*Decoder) RemainedRows ¶
RemainedRows indicates Decoder.intermChk has remained rows.
func (*Decoder) Reset ¶
Reset decodes data and store the result in Decoder.intermChk. This decode phase uses pointer operations with less CPU and memory costs.
func (*Decoder) ReuseIntermChk ¶
ReuseIntermChk swaps `Decoder.intermChk` with `chk` directly when `Decoder.intermChk.NumRows()` is no less than `chk.requiredRows * factor` where `factor` is 0.8 now. This can avoid the overhead of appending the data from `Decoder.intermChk` to `chk`. Moreover, the column.offsets needs to be further adjusted according to column.offset[0].
type DefaultColumnAllocator ¶
type DefaultColumnAllocator struct{}
DefaultColumnAllocator is the default implementation of ColumnAllocator.
type Iterator ¶
type Iterator interface { // Begin resets the cursor of the iterator and returns the first Row. Begin() Row // Next returns the next Row. Next() Row // End returns the invalid end Row. End() Row // Len returns the length. Len() int // Current returns the current Row. Current() Row // ReachEnd reaches the end of iterator. ReachEnd() // Error returns none-nil error if anything wrong happens during the iteration. Error() error }
Iterator is used to iterate a number of rows.
for row := it.Begin(); row != it.End(); row = it.Next() { ... }
func NewIterator4List ¶
NewIterator4List returns a Iterator for List.
func NewIterator4RowPtr ¶
NewIterator4RowPtr returns a Iterator for RowPtrs.
func NewMultiIterator ¶
NewMultiIterator creates a new multiIterator
type Iterator4Chunk ¶
type Iterator4Chunk struct {
// contains filtered or unexported fields
}
Iterator4Chunk is used to iterate rows inside a chunk.
func NewIterator4Chunk ¶
func NewIterator4Chunk(chk *Chunk) *Iterator4Chunk
NewIterator4Chunk returns a iterator for Chunk.
func (*Iterator4Chunk) Begin ¶
func (it *Iterator4Chunk) Begin() Row
Begin implements the Iterator interface.
func (*Iterator4Chunk) Current ¶
func (it *Iterator4Chunk) Current() Row
Current implements the Iterator interface.
func (*Iterator4Chunk) End ¶
func (*Iterator4Chunk) End() Row
End implements the Iterator interface.
func (*Iterator4Chunk) Error ¶
func (*Iterator4Chunk) Error() error
Error returns none-nil error if anything wrong happens during the iteration.
func (*Iterator4Chunk) GetChunk ¶
func (it *Iterator4Chunk) GetChunk() *Chunk
GetChunk returns the chunk stored in the Iterator4Chunk
func (*Iterator4Chunk) Len ¶
func (it *Iterator4Chunk) Len() int
Len implements the Iterator interface
func (*Iterator4Chunk) Next ¶
func (it *Iterator4Chunk) Next() Row
Next implements the Iterator interface.
func (*Iterator4Chunk) ReachEnd ¶
func (it *Iterator4Chunk) ReachEnd()
ReachEnd implements the Iterator interface.
func (*Iterator4Chunk) ResetChunk ¶
func (it *Iterator4Chunk) ResetChunk(chk *Chunk)
ResetChunk reset the chunk this iter holds
type Iterator4Slice ¶
type Iterator4Slice struct {
// contains filtered or unexported fields
}
Iterator4Slice is used to iterate rows inside a slice.
func NewIterator4Slice ¶
func NewIterator4Slice(rows []Row) *Iterator4Slice
NewIterator4Slice returns a *Iterator4Slice for Row slice.
func (*Iterator4Slice) Begin ¶
func (it *Iterator4Slice) Begin() Row
Begin implements the Iterator interface.
func (*Iterator4Slice) Current ¶
func (it *Iterator4Slice) Current() Row
Current implements the Iterator interface.
func (*Iterator4Slice) End ¶
func (*Iterator4Slice) End() Row
End implements the Iterator interface.
func (*Iterator4Slice) Error ¶
func (*Iterator4Slice) Error() error
Error returns none-nil error if anything wrong happens during the iteration.
func (*Iterator4Slice) Len ¶
func (it *Iterator4Slice) Len() int
Len implements the Iterator interface.
func (*Iterator4Slice) Next ¶
func (it *Iterator4Slice) Next() Row
Next implements the Iterator interface.
func (*Iterator4Slice) ReachEnd ¶
func (it *Iterator4Slice) ReachEnd()
ReachEnd implements the Iterator interface.
func (*Iterator4Slice) Reset ¶
func (it *Iterator4Slice) Reset(rows []Row)
Reset iterator.rows and cursor.
type List ¶
type List struct {
// contains filtered or unexported fields
}
List holds a slice of chunks, use to append rows with max chunk size properly handled.
func NewListWithMemTracker ¶
func NewListWithMemTracker(fieldTypes []*types.FieldType, initChunkSize, maxChunkSize int, tracker *memory.Tracker) *List
NewListWithMemTracker creates a new List with field types, init chunk size, max chunk size and memory tracker.
func (*List) Add ¶
Add adds a chunk to the List, the chunk may be modified later by the list. Caller must make sure the input chk is not empty and not used any more and has the same field types.
func (*List) Clear ¶
func (l *List) Clear()
Clear triggers GC for all the allocated chunks and reset the list
func (*List) FieldTypes ¶
FieldTypes returns the fieldTypes of the list
func (*List) GetMemTracker ¶
GetMemTracker returns the memory tracker of this List.
func (*List) NumRowsOfChunk ¶
NumRowsOfChunk returns the number of rows of a chunk in the DataInDiskByRows.
func (*List) Walk ¶
func (l *List) Walk(walkFunc ListWalkFunc) error
Walk iterate the list and call walkFunc for each row.
type ListWalkFunc ¶
ListWalkFunc is used to walk the list. If error is returned, it will stop walking.
type MutRow ¶
type MutRow Row
MutRow represents a mutable Row. The underlying columns only contains one row and not exposed to the user.
func MutRowFromDatums ¶
MutRowFromDatums creates a MutRow from a datum slice.
func MutRowFromTypes ¶
MutRowFromTypes creates a MutRow from a FieldType slice, each Column is initialized to zero value.
func MutRowFromValues ¶
MutRowFromValues creates a MutRow from a interface slice.
func (MutRow) ShallowCopyPartialRow ¶
ShallowCopyPartialRow shallow copies the data of `row` to MutRow.
type Pool ¶
type Pool struct {
// contains filtered or unexported fields
}
Pool is the Column pool. NOTE: Pool is non-copyable.
type ReaderWithCache ¶
type ReaderWithCache struct {
// contains filtered or unexported fields
}
ReaderWithCache helps to read data that has not be flushed to underlying layer. By using ReaderWithCache, user can still write data into DataInDiskByRows even after reading.
func NewReaderWithCache ¶
func NewReaderWithCache(r io.ReaderAt, cache []byte, cacheOff int64) *ReaderWithCache
NewReaderWithCache returns a ReaderWithCache.
type Row ¶
type Row struct {
// contains filtered or unexported fields
}
Row represents a row of data, can be used to access values.
func (Row) CopyConstruct ¶
CopyConstruct creates a new row and copies this row's data into it.
func (Row) DatumWithBuffer ¶
DatumWithBuffer gets datum using the buffer d.
func (Row) GetDatumRow ¶
GetDatumRow converts chunk.Row to types.DatumRow. Keep in mind that GetDatumRow has a reference to r.c, which is a chunk, this function works only if the underlying chunk is valid or unchanged.
func (Row) GetDatumRowWithBuffer ¶
GetDatumRowWithBuffer gets datum using the buffer datumRow.
func (Row) GetDuration ¶
GetDuration returns the Duration value with the colIdx.
func (Row) GetFloat32 ¶
GetFloat32 returns the float32 value with the colIdx.
func (Row) GetFloat64 ¶
GetFloat64 returns the float64 value with the colIdx.
func (Row) GetJSON ¶
func (r Row) GetJSON(colIdx int) types.BinaryJSON
GetJSON returns the JSON value with the colIdx.
func (Row) GetMyDecimal ¶
GetMyDecimal returns the MyDecimal value with the colIdx.
func (Row) GetVectorFloat32 ¶
func (r Row) GetVectorFloat32(colIdx int) types.VectorFloat32
GetVectorFloat32 returns the VectorFloat32 value with the colIdx.
type RowContainer ¶
type RowContainer struct {
// contains filtered or unexported fields
}
RowContainer provides a place for many rows, so many that we might want to spill them into disk. nolint:structcheck
func NewRowContainer ¶
func NewRowContainer(fieldType []*types.FieldType, chunkSize int) *RowContainer
NewRowContainer creates a new RowContainer in memory.
func (*RowContainer) ActionSpill ¶
func (c *RowContainer) ActionSpill() *SpillDiskAction
ActionSpill returns a SpillDiskAction for spilling over to disk.
func (*RowContainer) ActionSpillForTest ¶
func (c *RowContainer) ActionSpillForTest() *SpillDiskAction
ActionSpillForTest returns a SpillDiskAction for spilling over to disk for test.
func (*RowContainer) Add ¶
func (c *RowContainer) Add(chk *Chunk) (err error)
Add appends a chunk into the RowContainer.
func (*RowContainer) AllocChunk ¶
func (c *RowContainer) AllocChunk() (chk *Chunk)
AllocChunk allocates a new chunk from RowContainer.
func (*RowContainer) AlreadySpilledSafeForTest ¶
func (c *RowContainer) AlreadySpilledSafeForTest() bool
AlreadySpilledSafeForTest indicates that records have spilled out into disk. It's thread-safe. The function is only used for test.
func (*RowContainer) Close ¶
func (c *RowContainer) Close() (err error)
Close close the RowContainer
func (*RowContainer) GetChunk ¶
func (c *RowContainer) GetChunk(chkIdx int) (*Chunk, error)
GetChunk returns chkIdx th chunk of in memory records.
func (*RowContainer) GetDiskTracker ¶
func (c *RowContainer) GetDiskTracker() *disk.Tracker
GetDiskTracker returns the underlying disk usage tracker in recordsInDisk.
func (*RowContainer) GetMemTracker ¶
func (c *RowContainer) GetMemTracker() *memory.Tracker
GetMemTracker returns the memory tracker in records, panics if the RowContainer has already spilled.
func (*RowContainer) GetRow ¶
func (c *RowContainer) GetRow(ptr RowPtr) (row Row, err error)
GetRow returns the row the ptr pointed to.
func (*RowContainer) GetRowAndAlwaysAppendToChunk ¶
func (c *RowContainer) GetRowAndAlwaysAppendToChunk(ptr RowPtr, chk *Chunk) (row Row, _ *Chunk, err error)
GetRowAndAlwaysAppendToChunk gets a Row from the RowContainer by RowPtr. Unlike `GetRowAndAppendToChunkIfInDisk`, this function always appends the row to the chunk, without considering whether it has spilled. It'll return `nil` chunk if it returns an error, or the chunk will be the same with the argument.
func (*RowContainer) GetRowAndAppendToChunkIfInDisk ¶
func (c *RowContainer) GetRowAndAppendToChunkIfInDisk(ptr RowPtr, chk *Chunk) (row Row, _ *Chunk, err error)
GetRowAndAppendToChunkIfInDisk gets a Row from the RowContainer by RowPtr. If the container has spilled, the row will be appended to the chunk. It'll return `nil` chunk if the container hasn't spilled, or it returns an error.
func (*RowContainer) NumChunks ¶
func (c *RowContainer) NumChunks() int
NumChunks returns the number of chunks in the container.
func (*RowContainer) NumRow ¶
func (c *RowContainer) NumRow() int
NumRow returns the number of rows in the container
func (*RowContainer) NumRowsOfChunk ¶
func (c *RowContainer) NumRowsOfChunk(chkID int) int
NumRowsOfChunk returns the number of rows of a chunk in the DataInDiskByRows.
func (*RowContainer) ShallowCopyWithNewMutex ¶
func (c *RowContainer) ShallowCopyWithNewMutex() *RowContainer
ShallowCopyWithNewMutex shallow clones a RowContainer. The new RowContainer shares the same underlying data with the old one but holds an individual rLock.
func (*RowContainer) SpillToDisk ¶
func (c *RowContainer) SpillToDisk()
SpillToDisk spills data to disk. This function may be called in parallel.
type RowContainerReader ¶
type RowContainerReader interface { // Next returns the next Row. Next() Row // Current returns the current Row. Current() Row // End returns the invalid end Row. End() Row // Error returns none-nil error if anything wrong happens during the iteration. Error() error // Close closes the dumper Close() }
RowContainerReader is a forward-only iterator for the row container. It provides an interface similar to other iterators, but it doesn't provide `ReachEnd` function and requires manually closing to release goroutine.
It's recommended to use the following pattern to use it:
for iter := NewRowContainerReader(rc); iter.Current() != iter.End(); iter.Next() { ... } iter.Close() if iter.Error() != nil { }
type RowPtr ¶
RowPtr is used to get a row from a list. It is only valid for the list that returns it.
type SortAndSpillDiskAction ¶
type SortAndSpillDiskAction struct {
// contains filtered or unexported fields
}
SortAndSpillDiskAction implements memory.ActionOnExceed for chunk.List. If the memory quota of a query is exceeded, SortAndSpillDiskAction.Action is triggered.
func (*SortAndSpillDiskAction) Action ¶
func (a *SortAndSpillDiskAction) Action(t *memory.Tracker)
Action sends a signal to trigger sortAndSpillToDisk method of RowContainer and if it is already triggered before, call its fallbackAction.
func (SortAndSpillDiskAction) GetPriority ¶
func (SortAndSpillDiskAction) GetPriority() int64
GetPriority get the priority of the Action.
func (SortAndSpillDiskAction) Reset ¶
func (a SortAndSpillDiskAction) Reset()
Reset resets the status for SpillDiskAction.
func (*SortAndSpillDiskAction) WaitForTest ¶
func (a *SortAndSpillDiskAction) WaitForTest()
WaitForTest waits all goroutine have gone.
type SortedRowContainer ¶
type SortedRowContainer struct { *RowContainer ByItemsDesc []bool // contains filtered or unexported fields }
SortedRowContainer provides a place for many rows, so many that we might want to sort and spill them into disk.
func NewSortedRowContainer ¶
func NewSortedRowContainer(fieldType []*types.FieldType, chunkSize int, byItemsDesc []bool, keyColumns []int, keyCmpFuncs []CompareFunc) *SortedRowContainer
NewSortedRowContainer creates a new SortedRowContainer in memory.
func (*SortedRowContainer) ActionSpill ¶
func (c *SortedRowContainer) ActionSpill() *SortAndSpillDiskAction
ActionSpill returns a SortAndSpillDiskAction for sorting and spilling over to disk.
func (*SortedRowContainer) ActionSpillForTest ¶
func (c *SortedRowContainer) ActionSpillForTest() *SortAndSpillDiskAction
ActionSpillForTest returns a SortAndSpillDiskAction for sorting and spilling over to disk for test.
func (*SortedRowContainer) Add ¶
func (c *SortedRowContainer) Add(chk *Chunk) (err error)
Add appends a chunk into the SortedRowContainer.
func (*SortedRowContainer) Close ¶
func (c *SortedRowContainer) Close() error
Close close the SortedRowContainer
func (*SortedRowContainer) GetMemTracker ¶
func (c *SortedRowContainer) GetMemTracker() *memory.Tracker
GetMemTracker return the memory tracker for the sortedRowContainer
func (*SortedRowContainer) GetSortedRow ¶
func (c *SortedRowContainer) GetSortedRow(idx int) (Row, error)
GetSortedRow returns the row the idx pointed to.
func (*SortedRowContainer) GetSortedRowAndAlwaysAppendToChunk ¶
func (c *SortedRowContainer) GetSortedRowAndAlwaysAppendToChunk(idx int, chk *Chunk) (Row, *Chunk, error)
GetSortedRowAndAlwaysAppendToChunk returns the row the idx pointed to.
func (*SortedRowContainer) Sort ¶
func (c *SortedRowContainer) Sort() (ret error)
Sort inits pointers and sorts the records.
func (*SortedRowContainer) SpillToDisk ¶
func (c *SortedRowContainer) SpillToDisk()
SpillToDisk spills data to disk. This function may be called in parallel.
type SpillDiskAction ¶
type SpillDiskAction struct {
// contains filtered or unexported fields
}
SpillDiskAction implements memory.ActionOnExceed for chunk.List. If the memory quota of a query is exceeded, SpillDiskAction.Action is triggered.
func (*SpillDiskAction) Action ¶
func (a *SpillDiskAction) Action(t *memory.Tracker)
Action sends a signal to trigger spillToDisk method of RowContainer and if it is already triggered before, call its fallbackAction.
func (SpillDiskAction) GetPriority ¶
func (SpillDiskAction) GetPriority() int64
GetPriority get the priority of the Action.
func (SpillDiskAction) Reset ¶
func (a SpillDiskAction) Reset()
Reset resets the status for SpillDiskAction.
func (SpillDiskAction) WaitForTest ¶
func (a SpillDiskAction) WaitForTest()
WaitForTest waits all goroutine have gone.