-
Notifications
You must be signed in to change notification settings - Fork 454
New issue
Have a question about this project? Sign up for a free GitHub account to open an issue and contact its maintainers and the community.
By clicking “Sign up for GitHub”, you agree to our terms of service and privacy statement. We’ll occasionally send you account related emails.
Already on GitHub? Sign in to your account
[dbnode] Concurrent time series indexing within a single batch #2146
Changes from 8 commits
dbaeab1
9662f7a
c93ab02
81692ed
8be4138
97d56ff
f8a84d8
77ffa29
6eab81a
bbb3a84
ac934c2
File filter
Filter by extension
Conversations
Jump to
Diff view
Diff view
There are no files selected for viewing
Original file line number | Diff line number | Diff line change |
---|---|---|
|
@@ -23,18 +23,43 @@ package builder | |
import ( | ||
"errors" | ||
"fmt" | ||
"sync" | ||
|
||
"github.com/m3db/m3/src/m3ninx/doc" | ||
"github.com/m3db/m3/src/m3ninx/index" | ||
"github.com/m3db/m3/src/m3ninx/index/segment" | ||
"github.com/m3db/m3/src/m3ninx/postings" | ||
"github.com/m3db/m3/src/m3ninx/util" | ||
|
||
"github.com/cespare/xxhash" | ||
) | ||
|
||
var ( | ||
errDocNotFound = errors.New("doc not found") | ||
errClosed = errors.New("builder closed") | ||
) | ||
|
||
const ( | ||
// Slightly buffer the work to avoid blocking main thread. | ||
indexQueueSize = 2 << 9 // 1024 | ||
) | ||
|
||
type indexJob struct { | ||
wg *sync.WaitGroup | ||
|
||
id postings.ID | ||
field doc.Field | ||
|
||
shard int | ||
idx int | ||
batchErr *index.BatchPartialError | ||
} | ||
|
||
type builderStatus struct { | ||
sync.RWMutex | ||
closed bool | ||
} | ||
|
||
type builder struct { | ||
opts Options | ||
newUUIDFn util.NewUUIDFn | ||
|
@@ -44,29 +69,47 @@ type builder struct { | |
batchSizeOne index.Batch | ||
docs []doc.Document | ||
idSet *IDsMap | ||
fields *fieldsMap | ||
uniqueFields [][]byte | ||
fields *shardedFieldsMap | ||
uniqueFields [][][]byte | ||
|
||
indexQueues []chan indexJob | ||
status builderStatus | ||
} | ||
|
||
// NewBuilderFromDocuments returns a builder from documents, it is | ||
// not thread safe and is optimized for insertion speed and a | ||
// final build step when documents are indexed. | ||
func NewBuilderFromDocuments(opts Options) (segment.DocumentsBuilder, error) { | ||
return &builder{ | ||
func NewBuilderFromDocuments(opts Options) (segment.CloseableDocumentsBuilder, error) { | ||
concurrency := opts.Concurrency() | ||
b := &builder{ | ||
opts: opts, | ||
newUUIDFn: opts.NewUUIDFn(), | ||
batchSizeOne: index.Batch{ | ||
Docs: make([]doc.Document, 1), | ||
AllowPartialUpdates: false, | ||
Docs: make([]doc.Document, 1), | ||
}, | ||
idSet: NewIDsMap(IDsMapOptions{ | ||
InitialSize: opts.InitialCapacity(), | ||
}), | ||
fields: newFieldsMap(fieldsMapOptions{ | ||
InitialSize: opts.InitialCapacity(), | ||
}), | ||
uniqueFields: make([][]byte, 0, opts.InitialCapacity()), | ||
}, nil | ||
uniqueFields: make([][][]byte, 0, concurrency), | ||
indexQueues: make([]chan indexJob, 0, concurrency), | ||
} | ||
|
||
for i := 0; i < concurrency; i++ { | ||
indexQueue := make(chan indexJob, indexQueueSize) | ||
b.indexQueues = append(b.indexQueues, indexQueue) | ||
go b.indexWorker(indexQueue) | ||
robskillington marked this conversation as resolved.
Show resolved
Hide resolved
|
||
|
||
// Give each shard a fraction of the configured initial capacity. | ||
shardInitialCapcity := opts.InitialCapacity() | ||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more.
|
||
if shardInitialCapcity > 0 { | ||
shardInitialCapcity /= concurrency | ||
} | ||
shardUniqueFields := make([][]byte, 0, shardInitialCapcity) | ||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. This may be more performant if you allocate the entire shardUniqueFields := make([][]byte, 0, concurrency * shardInitialCapcity)
for i := 0; i < concurrency; i++ {
//...
b.uniqueFields = append(b.uniqueFields, shardUniqueFields[i*concurrency:(i+1)*concurrency])
// ...
} There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Wouldn't be able to grow a shard in this case. Or it would be difficult to do so. |
||
b.uniqueFields = append(b.uniqueFields, shardUniqueFields) | ||
b.fields = newShardedFieldsMap(concurrency, shardInitialCapcity) | ||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. May be possible to bulk-allocate these similarly to b.uniqueFields? There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Same comment as before. Growing a shard would be very painful. |
||
} | ||
|
||
return b, nil | ||
} | ||
|
||
func (b *builder) Reset(offset postings.ID) { | ||
|
@@ -83,15 +126,15 @@ func (b *builder) Reset(offset postings.ID) { | |
b.idSet.Reset() | ||
|
||
// Keep fields around, just reset the terms set for each one. | ||
for _, entry := range b.fields.Iter() { | ||
entry.Value().reset() | ||
} | ||
b.fields.ResetTermsSets() | ||
|
||
// Reset the unique fields slice | ||
for i := range b.uniqueFields { | ||
b.uniqueFields[i] = nil | ||
for i, shardUniqueFields := range b.uniqueFields { | ||
for i := range shardUniqueFields { | ||
shardUniqueFields[i] = nil | ||
} | ||
b.uniqueFields[i] = shardUniqueFields[:0] | ||
} | ||
b.uniqueFields = b.uniqueFields[:0] | ||
} | ||
|
||
func (b *builder) Insert(d doc.Document) ([]byte, error) { | ||
|
@@ -107,15 +150,20 @@ func (b *builder) Insert(d doc.Document) ([]byte, error) { | |
} | ||
|
||
func (b *builder) InsertBatch(batch index.Batch) error { | ||
b.status.RLock() | ||
defer b.status.RUnlock() | ||
|
||
if b.status.closed { | ||
return errClosed | ||
} | ||
|
||
// NB(r): This is all kept in a single method to make the | ||
// insertion path fast. | ||
var wg sync.WaitGroup | ||
batchErr := index.NewBatchPartialError() | ||
for i, d := range batch.Docs { | ||
// Validate doc | ||
if err := d.Validate(); err != nil { | ||
if !batch.AllowPartialUpdates { | ||
return err | ||
} | ||
batchErr.Add(index.BatchError{Err: err, Idx: i}) | ||
continue | ||
} | ||
|
@@ -124,9 +172,6 @@ func (b *builder) InsertBatch(batch index.Batch) error { | |
if !d.HasID() { | ||
id, err := b.newUUIDFn() | ||
if err != nil { | ||
if !batch.AllowPartialUpdates { | ||
return err | ||
} | ||
batchErr.Add(index.BatchError{Err: err, Idx: i}) | ||
continue | ||
} | ||
|
@@ -139,9 +184,6 @@ func (b *builder) InsertBatch(batch index.Batch) error { | |
|
||
// Avoid duplicates. | ||
if _, ok := b.idSet.Get(d.ID); ok { | ||
if !batch.AllowPartialUpdates { | ||
return index.ErrDuplicateID | ||
} | ||
batchErr.Add(index.BatchError{Err: index.ErrDuplicateID, Idx: i}) | ||
continue | ||
} | ||
|
@@ -158,50 +200,70 @@ func (b *builder) InsertBatch(batch index.Batch) error { | |
|
||
// Index the terms. | ||
for _, f := range d.Fields { | ||
if err := b.index(postings.ID(postingsListID), f); err != nil { | ||
if !batch.AllowPartialUpdates { | ||
return err | ||
} | ||
batchErr.Add(index.BatchError{Err: err, Idx: i}) | ||
} | ||
b.index(&wg, postings.ID(postingsListID), f, i, batchErr) | ||
} | ||
if err := b.index(postings.ID(postingsListID), doc.Field{ | ||
b.index(&wg, postings.ID(postingsListID), doc.Field{ | ||
Name: doc.IDReservedFieldName, | ||
Value: d.ID, | ||
}); err != nil { | ||
if !batch.AllowPartialUpdates { | ||
return err | ||
} | ||
batchErr.Add(index.BatchError{Err: err, Idx: i}) | ||
} | ||
}, i, batchErr) | ||
} | ||
|
||
// Wait for all the concurrent indexing jobs to finish. | ||
wg.Wait() | ||
|
||
if !batchErr.IsEmpty() { | ||
return batchErr | ||
} | ||
return nil | ||
} | ||
|
||
func (b *builder) index(id postings.ID, f doc.Field) error { | ||
terms, ok := b.fields.Get(f.Name) | ||
if !ok { | ||
terms = newTerms(b.opts) | ||
b.fields.SetUnsafe(f.Name, terms, fieldsMapSetUnsafeOptions{ | ||
NoCopyKey: true, | ||
NoFinalizeKey: true, | ||
}) | ||
func (b *builder) index( | ||
wg *sync.WaitGroup, | ||
id postings.ID, | ||
f doc.Field, | ||
i int, | ||
batchErr *index.BatchPartialError, | ||
) { | ||
wg.Add(1) | ||
// NB(bodu): To avoid locking inside of the terms, we shard the work | ||
// by field name. | ||
shard := int(xxhash.Sum64(f.Name) % uint64(len(b.indexQueues))) | ||
b.indexQueues[shard] <- indexJob{ | ||
wg: wg, | ||
id: id, | ||
field: f, | ||
shard: shard, | ||
idx: i, | ||
batchErr: batchErr, | ||
} | ||
} | ||
|
||
// If empty field, track insertion of this key into the fields | ||
// collection for correct response when retrieving all fields. | ||
newField := terms.size() == 0 | ||
if err := terms.post(f.Value, id); err != nil { | ||
return err | ||
} | ||
if newField { | ||
b.uniqueFields = append(b.uniqueFields, f.Name) | ||
func (b *builder) indexWorker(indexQueue chan indexJob) { | ||
for job := range indexQueue { | ||
terms, ok := b.fields.ShardedGet(job.shard, job.field.Name) | ||
if !ok { | ||
// NB(bodu): Check again within the lock to make sure we aren't making concurrent map writes. | ||
terms = newTerms(b.opts) | ||
b.fields.ShardedSetUnsafe(job.shard, job.field.Name, terms, fieldsMapSetUnsafeOptions{ | ||
NoCopyKey: true, | ||
NoFinalizeKey: true, | ||
}) | ||
} | ||
|
||
// If empty field, track insertion of this key into the fields | ||
// collection for correct response when retrieving all fields. | ||
newField := terms.size() == 0 | ||
// NB(bodu): Bulk of the cpu time during insertion is spent inside of terms.post(). | ||
err := terms.post(job.field.Value, job.id) | ||
if err == nil { | ||
if newField { | ||
b.uniqueFields[job.shard] = append(b.uniqueFields[job.shard], job.field.Name) | ||
} | ||
} else { | ||
job.batchErr.AddWithLock(index.BatchError{Err: err, Idx: job.idx}) | ||
} | ||
notbdu marked this conversation as resolved.
Show resolved
Hide resolved
|
||
job.wg.Done() | ||
} | ||
return nil | ||
} | ||
|
||
func (b *builder) AllDocs() (index.IDDocIterator, error) { | ||
|
@@ -236,7 +298,9 @@ func (b *builder) Fields() (segment.FieldsIterator, error) { | |
} | ||
|
||
func (b *builder) Terms(field []byte) (segment.TermsIterator, error) { | ||
terms, ok := b.fields.Get(field) | ||
// NB(bodu): The # of indexQueues and field map shards are equal. | ||
shard := int(xxhash.Sum64(field) % uint64(len(b.indexQueues))) | ||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Can we make this a method on the builder to reuse this code? I see it replicated on this line and line 230 i.e. func (b *builder) shardForField(field []byte) {
return int(xxhash.Sum64(field) % uint64(len(b.indexQueues)))
} |
||
terms, ok := b.fields.ShardedGet(shard, field) | ||
if !ok { | ||
return nil, fmt.Errorf("field not found: %s", string(field)) | ||
} | ||
|
@@ -247,3 +311,13 @@ func (b *builder) Terms(field []byte) (segment.TermsIterator, error) { | |
|
||
return newTermsIter(terms.uniqueTerms), nil | ||
} | ||
|
||
func (b *builder) Close() error { | ||
b.status.Lock() | ||
defer b.status.Unlock() | ||
for _, q := range b.indexQueues { | ||
close(q) | ||
robskillington marked this conversation as resolved.
Show resolved
Hide resolved
|
||
} | ||
b.status.closed = true | ||
return nil | ||
} |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
Good catch.