-
Notifications
You must be signed in to change notification settings - Fork 453
/
segment.go
634 lines (536 loc) · 15.7 KB
/
segment.go
1
2
3
4
5
6
7
8
9
10
11
12
13
14
15
16
17
18
19
20
21
22
23
24
25
26
27
28
29
30
31
32
33
34
35
36
37
38
39
40
41
42
43
44
45
46
47
48
49
50
51
52
53
54
55
56
57
58
59
60
61
62
63
64
65
66
67
68
69
70
71
72
73
74
75
76
77
78
79
80
81
82
83
84
85
86
87
88
89
90
91
92
93
94
95
96
97
98
99
100
101
102
103
104
105
106
107
108
109
110
111
112
113
114
115
116
117
118
119
120
121
122
123
124
125
126
127
128
129
130
131
132
133
134
135
136
137
138
139
140
141
142
143
144
145
146
147
148
149
150
151
152
153
154
155
156
157
158
159
160
161
162
163
164
165
166
167
168
169
170
171
172
173
174
175
176
177
178
179
180
181
182
183
184
185
186
187
188
189
190
191
192
193
194
195
196
197
198
199
200
201
202
203
204
205
206
207
208
209
210
211
212
213
214
215
216
217
218
219
220
221
222
223
224
225
226
227
228
229
230
231
232
233
234
235
236
237
238
239
240
241
242
243
244
245
246
247
248
249
250
251
252
253
254
255
256
257
258
259
260
261
262
263
264
265
266
267
268
269
270
271
272
273
274
275
276
277
278
279
280
281
282
283
284
285
286
287
288
289
290
291
292
293
294
295
296
297
298
299
300
301
302
303
304
305
306
307
308
309
310
311
312
313
314
315
316
317
318
319
320
321
322
323
324
325
326
327
328
329
330
331
332
333
334
335
336
337
338
339
340
341
342
343
344
345
346
347
348
349
350
351
352
353
354
355
356
357
358
359
360
361
362
363
364
365
366
367
368
369
370
371
372
373
374
375
376
377
378
379
380
381
382
383
384
385
386
387
388
389
390
391
392
393
394
395
396
397
398
399
400
401
402
403
404
405
406
407
408
409
410
411
412
413
414
415
416
417
418
419
420
421
422
423
424
425
426
427
428
429
430
431
432
433
434
435
436
437
438
439
440
441
442
443
444
445
446
447
448
449
450
451
452
453
454
455
456
457
458
459
460
461
462
463
464
465
466
467
468
469
470
471
472
473
474
475
476
477
478
479
480
481
482
483
484
485
486
487
488
489
490
491
492
493
494
495
496
497
498
499
500
501
502
503
504
505
506
507
508
509
510
511
512
513
514
515
516
517
518
519
520
521
522
523
524
525
526
527
528
529
530
531
532
533
534
535
536
537
538
539
540
541
542
543
544
545
546
547
548
549
550
551
552
553
554
555
556
557
558
559
560
561
562
563
564
565
566
567
568
569
570
571
572
573
574
575
576
577
578
579
580
581
582
583
584
585
586
587
588
589
590
591
592
593
594
595
596
597
598
599
600
601
602
603
604
605
606
607
608
609
610
611
612
613
614
615
616
617
618
619
620
621
622
623
624
625
626
627
628
629
630
631
632
633
634
// Copyright (c) 2018 Uber Technologies, Inc.
//
// Permission is hereby granted, free of charge, to any person obtaining a copy
// of this software and associated documentation files (the "Software"), to deal
// in the Software without restriction, including without limitation the rights
// to use, copy, modify, merge, publish, distribute, sublicense, and/or sell
// copies of the Software, and to permit persons to whom the Software is
// furnished to do so, subject to the following conditions:
//
// The above copyright notice and this permission notice shall be included in
// all copies or substantial portions of the Software.
//
// THE SOFTWARE IS PROVIDED "AS IS", WITHOUT WARRANTY OF ANY KIND, EXPRESS OR
// IMPLIED, INCLUDING BUT NOT LIMITED TO THE WARRANTIES OF MERCHANTABILITY,
// FITNESS FOR A PARTICULAR PURPOSE AND NONINFRINGEMENT. IN NO EVENT SHALL THE
// AUTHORS OR COPYRIGHT HOLDERS BE LIABLE FOR ANY CLAIM, DAMAGES OR OTHER
// LIABILITY, WHETHER IN AN ACTION OF CONTRACT, TORT OR OTHERWISE, ARISING FROM,
// OUT OF OR IN CONNECTION WITH THE SOFTWARE OR THE USE OR OTHER DEALINGS IN
// THE SOFTWARE.
package fst
import (
"errors"
"fmt"
"io"
"sync"
"github.com/m3db/m3/src/m3ninx/doc"
"github.com/m3db/m3/src/m3ninx/generated/proto/fswriter"
"github.com/m3db/m3/src/m3ninx/index"
sgmt "github.com/m3db/m3/src/m3ninx/index/segment"
"github.com/m3db/m3/src/m3ninx/index/segment/fst/encoding"
"github.com/m3db/m3/src/m3ninx/index/segment/fst/encoding/docs"
"github.com/m3db/m3/src/m3ninx/postings"
"github.com/m3db/m3/src/m3ninx/postings/pilosa"
"github.com/m3db/m3/src/m3ninx/postings/roaring"
"github.com/m3db/m3/src/m3ninx/x"
xerrors "github.com/m3db/m3x/errors"
pilosaroaring "github.com/m3db/pilosa/roaring"
"github.com/couchbase/vellum"
)
var (
errReaderClosed = errors.New("segment is closed")
errReaderNilRegexp = errors.New("nil regexp provided")
errUnsupportedMajorVersion = errors.New("unsupported major version")
errDocumentsDataUnset = errors.New("documents data bytes are not set")
errDocumentsIdxUnset = errors.New("documents index bytes are not set")
errPostingsDataUnset = errors.New("postings data bytes are not set")
errFSTTermsDataUnset = errors.New("fst terms data bytes are not set")
errFSTFieldsDataUnset = errors.New("fst fields data bytes are not set")
)
// SegmentData represent the collection of required parameters to construct a Segment.
type SegmentData struct {
MajorVersion int
MinorVersion int
Metadata []byte
DocsData []byte
DocsIdxData []byte
PostingsData []byte
FSTTermsData []byte
FSTFieldsData []byte
// DocsReader is an alternative to specifying
// the docs data and docs idx data if the documents
// already reside in memory and we want to use the
// in memory references instead.
DocsReader *docs.SliceReader
Closer io.Closer
}
// Validate validates the provided segment data, returning an error if it's not.
func (sd SegmentData) Validate() error {
if sd.MajorVersion != MajorVersion {
return errUnsupportedMajorVersion
}
if sd.PostingsData == nil {
return errPostingsDataUnset
}
if sd.FSTTermsData == nil {
return errFSTTermsDataUnset
}
if sd.FSTFieldsData == nil {
return errFSTFieldsDataUnset
}
if sd.DocsReader == nil {
if sd.DocsData == nil {
return errDocumentsDataUnset
}
if sd.DocsIdxData == nil {
return errDocumentsIdxUnset
}
}
return nil
}
// NewSegment returns a new Segment backed by the provided options.
// NB(prateek): this method only assumes ownership of the data if it returns a nil error,
// otherwise, the user is expected to handle the lifecycle of the input.
func NewSegment(data SegmentData, opts Options) (Segment, error) {
if err := data.Validate(); err != nil {
return nil, err
}
metadata := fswriter.Metadata{}
if err := metadata.Unmarshal(data.Metadata); err != nil {
return nil, err
}
if metadata.PostingsFormat != fswriter.PostingsFormat_PILOSAV1_POSTINGS_FORMAT {
return nil, fmt.Errorf("unsupported postings format: %v", metadata.PostingsFormat.String())
}
fieldsFST, err := vellum.Load(data.FSTFieldsData)
if err != nil {
return nil, fmt.Errorf("unable to load fields fst: %v", err)
}
var (
docsSliceReader = data.DocsReader
docsDataReader *docs.DataReader
docsIndexReader *docs.IndexReader
startInclusive postings.ID
endExclusive postings.ID
)
if docsSliceReader != nil {
startInclusive = docsSliceReader.Base()
endExclusive = startInclusive + postings.ID(docsSliceReader.Len())
} else {
docsDataReader = docs.NewDataReader(data.DocsData)
docsIndexReader, err = docs.NewIndexReader(data.DocsIdxData)
if err != nil {
return nil, fmt.Errorf("unable to load documents index: %v", err)
}
// NB(jeromefroe): Currently we assume the postings IDs are contiguous.
startInclusive = docsIndexReader.Base()
endExclusive = startInclusive + postings.ID(docsIndexReader.Len())
}
return &fsSegment{
fieldsFST: fieldsFST,
docsDataReader: docsDataReader,
docsIndexReader: docsIndexReader,
docsSliceReader: docsSliceReader,
data: data,
opts: opts,
numDocs: metadata.NumDocs,
startInclusive: startInclusive,
endExclusive: endExclusive,
}, nil
}
type fsSegment struct {
sync.RWMutex
closed bool
fieldsFST *vellum.FST
docsDataReader *docs.DataReader
docsIndexReader *docs.IndexReader
docsSliceReader *docs.SliceReader
data SegmentData
opts Options
numDocs int64
startInclusive postings.ID
endExclusive postings.ID
}
func (r *fsSegment) Size() int64 {
r.RLock()
defer r.RUnlock()
if r.closed {
return 0
}
return r.numDocs
}
func (r *fsSegment) ContainsID(docID []byte) (bool, error) {
r.RLock()
defer r.RUnlock()
if r.closed {
return false, errReaderClosed
}
termsFST, exists, err := r.retrieveTermsFSTWithRLock(doc.IDReservedFieldName)
if err != nil {
return false, err
}
if !exists {
return false, fmt.Errorf("internal error while retrieving id FST: %v", err)
}
_, exists, err = termsFST.Get(docID)
closeErr := termsFST.Close()
if err != nil {
return false, err
}
return exists, closeErr
}
func (r *fsSegment) Reader() (index.Reader, error) {
r.RLock()
defer r.RUnlock()
if r.closed {
return nil, errReaderClosed
}
return &fsSegmentReader{
fsSegment: r,
}, nil
}
func (r *fsSegment) Close() error {
r.Lock()
defer r.Unlock()
if r.closed {
return errReaderClosed
}
r.closed = true
var multiErr xerrors.MultiError
multiErr = multiErr.Add(r.fieldsFST.Close())
if r.data.Closer != nil {
multiErr = multiErr.Add(r.data.Closer.Close())
}
return multiErr.FinalError()
}
func (r *fsSegment) FieldsIterable() sgmt.FieldsIterable {
return r
}
func (r *fsSegment) Fields() (sgmt.FieldsIterator, error) {
r.RLock()
defer r.RUnlock()
if r.closed {
return nil, errReaderClosed
}
iter := newFSTTermsIter()
iter.reset(fstTermsIterOpts{
fst: r.fieldsFST,
finalizeFST: false,
})
return iter, nil
}
func (r *fsSegment) TermsIterable() sgmt.TermsIterable {
return &termsIterable{
r: r,
fieldsIter: newFSTTermsIter(),
postingsIter: newFSTTermsPostingsIter(),
}
}
// termsIterable allows multiple term lookups to share the same roaring
// bitmap being unpacked for use when iterating over an entire segment
type termsIterable struct {
r *fsSegment
fieldsIter *fstTermsIter
postingsIter *fstTermsPostingsIter
}
func (i *termsIterable) Terms(field []byte) (sgmt.TermsIterator, error) {
i.r.RLock()
defer i.r.RUnlock()
if i.r.closed {
return nil, errReaderClosed
}
termsFST, exists, err := i.r.retrieveTermsFSTWithRLock(field)
if err != nil {
return nil, err
}
if !exists {
return sgmt.EmptyTermsIterator, nil
}
i.fieldsIter.reset(fstTermsIterOpts{
fst: termsFST,
finalizeFST: true,
})
i.postingsIter.reset(i.r, i.fieldsIter)
return i.postingsIter, nil
}
func (r *fsSegment) UnmarshalPostingsListBitmap(b *pilosaroaring.Bitmap, offset uint64) error {
r.RLock()
defer r.RUnlock()
if r.closed {
return errReaderClosed
}
postingsBytes, err := r.retrieveBytesWithRLock(r.data.PostingsData, offset)
if err != nil {
return fmt.Errorf("unable to retrieve postings data: %v", err)
}
b.Reset()
return b.UnmarshalBinary(postingsBytes)
}
func (r *fsSegment) MatchTerm(field []byte, term []byte) (postings.List, error) {
r.RLock()
defer r.RUnlock()
if r.closed {
return nil, errReaderClosed
}
termsFST, exists, err := r.retrieveTermsFSTWithRLock(field)
if err != nil {
return nil, err
}
if !exists {
// i.e. we don't know anything about the field, so can early return an empty postings list
return r.opts.PostingsListPool().Get(), nil
}
fstCloser := x.NewSafeCloser(termsFST)
defer fstCloser.Close()
postingsOffset, exists, err := termsFST.Get(term)
if err != nil {
return nil, err
}
if !exists {
// i.e. we don't know anything about the term, so can early return an empty postings list
return r.opts.PostingsListPool().Get(), nil
}
pl, err := r.retrievePostingsListWithRLock(postingsOffset)
if err != nil {
return nil, err
}
if err := fstCloser.Close(); err != nil {
return nil, err
}
return pl, nil
}
func (r *fsSegment) MatchRegexp(field []byte, compiled index.CompiledRegex) (postings.List, error) {
r.RLock()
defer r.RUnlock()
if r.closed {
return nil, errReaderClosed
}
re := compiled.FST
if re == nil {
return nil, errReaderNilRegexp
}
termsFST, exists, err := r.retrieveTermsFSTWithRLock(field)
if err != nil {
return nil, err
}
if !exists {
// i.e. we don't know anything about the field, so can early return an empty postings list
return r.opts.PostingsListPool().Get(), nil
}
var (
fstCloser = x.NewSafeCloser(termsFST)
iter, iterErr = termsFST.Search(re, compiled.PrefixBegin, compiled.PrefixEnd)
iterCloser = x.NewSafeCloser(iter)
// NB(prateek): way quicker to union the PLs together at the end, rathen than one at a time.
pls []postings.List // TODO: pool this slice allocation
)
defer func() {
iterCloser.Close()
fstCloser.Close()
}()
for {
if iterErr == vellum.ErrIteratorDone {
break
}
if iterErr != nil {
return nil, iterErr
}
_, postingsOffset := iter.Current()
nextPl, err := r.retrievePostingsListWithRLock(postingsOffset)
if err != nil {
return nil, err
}
pls = append(pls, nextPl)
iterErr = iter.Next()
}
pl, err := roaring.Union(pls)
if err != nil {
return nil, err
}
if err := iterCloser.Close(); err != nil {
return nil, err
}
if err := fstCloser.Close(); err != nil {
return nil, err
}
return pl, nil
}
func (r *fsSegment) MatchAll() (postings.MutableList, error) {
r.RLock()
defer r.RUnlock()
if r.closed {
return nil, errReaderClosed
}
pl := r.opts.PostingsListPool().Get()
err := pl.AddRange(r.startInclusive, r.endExclusive)
if err != nil {
return nil, err
}
return pl, nil
}
func (r *fsSegment) Doc(id postings.ID) (doc.Document, error) {
r.RLock()
defer r.RUnlock()
if r.closed {
return doc.Document{}, errReaderClosed
}
// If using docs slice reader, return from the in memory slice reader
if r.docsSliceReader != nil {
return r.docsSliceReader.Read(id)
}
offset, err := r.docsIndexReader.Read(id)
if err != nil {
return doc.Document{}, err
}
return r.docsDataReader.Read(offset)
}
func (r *fsSegment) Docs(pl postings.List) (doc.Iterator, error) {
r.RLock()
defer r.RUnlock()
if r.closed {
return nil, errReaderClosed
}
return index.NewIDDocIterator(r, pl.Iterator()), nil
}
func (r *fsSegment) AllDocs() (index.IDDocIterator, error) {
r.RLock()
defer r.RUnlock()
if r.closed {
return nil, errReaderClosed
}
pi := postings.NewRangeIterator(r.startInclusive, r.endExclusive)
return index.NewIDDocIterator(r, pi), nil
}
func (r *fsSegment) retrievePostingsListWithRLock(postingsOffset uint64) (postings.List, error) {
postingsBytes, err := r.retrieveBytesWithRLock(r.data.PostingsData, postingsOffset)
if err != nil {
return nil, fmt.Errorf("unable to retrieve postings data: %v", err)
}
return pilosa.Unmarshal(postingsBytes)
}
func (r *fsSegment) retrieveTermsFSTWithRLock(field []byte) (*vellum.FST, bool, error) {
termsFSTOffset, exists, err := r.fieldsFST.Get(field)
if err != nil {
return nil, false, err
}
if !exists {
return nil, false, nil
}
termsFSTBytes, err := r.retrieveBytesWithRLock(r.data.FSTTermsData, termsFSTOffset)
if err != nil {
return nil, false, fmt.Errorf("error while decoding terms fst: %v", err)
}
termsFST, err := vellum.Load(termsFSTBytes)
if err != nil {
return nil, false, fmt.Errorf("error while loading terms fst: %v", err)
}
return termsFST, true, nil
}
// retrieveBytesWithRLock assumes the base []byte slice is a collection of (payload, size, magicNumber) triples,
// where size/magicNumber are strictly uint64 (i.e. 8 bytes). It assumes the 8 bytes preceding the offset
// are the magicNumber, the 8 bytes before that are the size, and the `size` bytes before that are the
// payload. It retrieves the payload while doing bounds checks to ensure no segfaults.
func (r *fsSegment) retrieveBytesWithRLock(base []byte, offset uint64) ([]byte, error) {
const sizeofUint64 = 8
var (
magicNumberEnd = int64(offset) // to prevent underflows
magicNumberStart = offset - sizeofUint64
)
if magicNumberEnd > int64(len(base)) || magicNumberStart < 0 {
return nil, fmt.Errorf("base bytes too small, length: %d, base-offset: %d", len(base), magicNumberEnd)
}
magicNumberBytes := base[magicNumberStart:magicNumberEnd]
d := encoding.NewDecoder(magicNumberBytes)
n, err := d.Uint64()
if err != nil {
return nil, fmt.Errorf("error while decoding magicNumber: %v", err)
}
if n != uint64(magicNumber) {
return nil, fmt.Errorf("mismatch while decoding magicNumber: %d", n)
}
var (
sizeEnd = magicNumberStart
sizeStart = sizeEnd - sizeofUint64
)
if sizeStart < 0 {
return nil, fmt.Errorf("base bytes too small, length: %d, size-offset: %d", len(base), sizeStart)
}
sizeBytes := base[sizeStart:sizeEnd]
d.Reset(sizeBytes)
size, err := d.Uint64()
if err != nil {
return nil, fmt.Errorf("error while decoding size: %v", err)
}
var (
payloadEnd = sizeStart
payloadStart = payloadEnd - size
)
if payloadStart < 0 {
return nil, fmt.Errorf("base bytes too small, length: %d, payload-start: %d, payload-size: %d",
len(base), payloadStart, size)
}
return base[payloadStart:payloadEnd], nil
}
type fsSegmentReader struct {
sync.RWMutex
closed bool
fsSegment *fsSegment
}
var _ index.Reader = &fsSegmentReader{}
func (sr *fsSegmentReader) MatchTerm(field []byte, term []byte) (postings.List, error) {
sr.RLock()
defer sr.RUnlock()
if sr.closed {
return nil, errReaderClosed
}
return sr.fsSegment.MatchTerm(field, term)
}
func (sr *fsSegmentReader) MatchRegexp(field []byte, compiled index.CompiledRegex) (postings.List, error) {
sr.RLock()
defer sr.RUnlock()
if sr.closed {
return nil, errReaderClosed
}
return sr.fsSegment.MatchRegexp(field, compiled)
}
func (sr *fsSegmentReader) MatchAll() (postings.MutableList, error) {
sr.RLock()
defer sr.RUnlock()
if sr.closed {
return nil, errReaderClosed
}
return sr.fsSegment.MatchAll()
}
func (sr *fsSegmentReader) Doc(id postings.ID) (doc.Document, error) {
sr.RLock()
defer sr.RUnlock()
if sr.closed {
return doc.Document{}, errReaderClosed
}
return sr.fsSegment.Doc(id)
}
func (sr *fsSegmentReader) Docs(pl postings.List) (doc.Iterator, error) {
sr.RLock()
defer sr.RUnlock()
if sr.closed {
return nil, errReaderClosed
}
return sr.fsSegment.Docs(pl)
}
func (sr *fsSegmentReader) AllDocs() (index.IDDocIterator, error) {
sr.RLock()
defer sr.RUnlock()
if sr.closed {
return nil, errReaderClosed
}
return sr.fsSegment.AllDocs()
}
func (sr *fsSegmentReader) Close() error {
sr.Lock()
defer sr.Unlock()
if sr.closed {
return errReaderClosed
}
sr.closed = true
return nil
}