mirror of
https://github.com/k3s-io/kubernetes.git
synced 2025-07-19 18:02:01 +00:00
clusterip allocator based on IPAddress API
add a new ClusterIP allocator that uses the new IPAddress API resource and an informer as the backend, instead a bitmap snapshotted on etcd. Change-Id: Ia891a2900acd2682d4d169abab65cdd9270a8445
This commit is contained in:
parent
aa18a0cd3f
commit
b022475448
566
pkg/registry/core/service/ipallocator/ipallocator.go
Normal file
566
pkg/registry/core/service/ipallocator/ipallocator.go
Normal file
@ -0,0 +1,566 @@
|
||||
/*
|
||||
Copyright 2023 The Kubernetes Authors.
|
||||
|
||||
Licensed under the Apache License, Version 2.0 (the "License");
|
||||
you may not use this file except in compliance with the License.
|
||||
You may obtain a copy of the License at
|
||||
|
||||
http://www.apache.org/licenses/LICENSE-2.0
|
||||
|
||||
Unless required by applicable law or agreed to in writing, software
|
||||
distributed under the License is distributed on an "AS IS" BASIS,
|
||||
WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
|
||||
See the License for the specific language governing permissions and
|
||||
limitations under the License.
|
||||
*/
|
||||
|
||||
package ipallocator
|
||||
|
||||
import (
|
||||
"context"
|
||||
"fmt"
|
||||
"math"
|
||||
"math/big"
|
||||
"math/rand"
|
||||
"net"
|
||||
"net/netip"
|
||||
"time"
|
||||
|
||||
networkingv1alpha1 "k8s.io/api/networking/v1alpha1"
|
||||
apierrors "k8s.io/apimachinery/pkg/api/errors"
|
||||
metav1 "k8s.io/apimachinery/pkg/apis/meta/v1"
|
||||
"k8s.io/apimachinery/pkg/labels"
|
||||
networkingv1alpha1informers "k8s.io/client-go/informers/networking/v1alpha1"
|
||||
networkingv1alpha1client "k8s.io/client-go/kubernetes/typed/networking/v1alpha1"
|
||||
networkingv1alpha1listers "k8s.io/client-go/listers/networking/v1alpha1"
|
||||
"k8s.io/client-go/tools/cache"
|
||||
"k8s.io/klog/v2"
|
||||
api "k8s.io/kubernetes/pkg/apis/core"
|
||||
netutils "k8s.io/utils/net"
|
||||
utiltrace "k8s.io/utils/trace"
|
||||
)
|
||||
|
||||
const ControllerName = "ipallocator.k8s.io"
|
||||
|
||||
// Allocator implements current ipallocator interface using IPAddress API object
|
||||
// and an informer as backend.
|
||||
type Allocator struct {
|
||||
cidr *net.IPNet
|
||||
prefix netip.Prefix
|
||||
firstAddress netip.Addr // first IP address within the range
|
||||
offsetAddress netip.Addr // IP address that delimits the upper and lower subranges
|
||||
lastAddress netip.Addr // last IP address within the range
|
||||
family api.IPFamily // family is the IP family of this range
|
||||
|
||||
rangeOffset int // subdivides the assigned IP range to prefer dynamic allocation from the upper range
|
||||
size uint64 // cap the total number of IPs available to maxInt64
|
||||
|
||||
client networkingv1alpha1client.NetworkingV1alpha1Interface
|
||||
ipAddressLister networkingv1alpha1listers.IPAddressLister
|
||||
ipAddressSynced cache.InformerSynced
|
||||
|
||||
// metrics is a metrics recorder that can be disabled
|
||||
metrics metricsRecorderInterface
|
||||
metricLabel string
|
||||
|
||||
rand *rand.Rand
|
||||
}
|
||||
|
||||
var _ Interface = &Allocator{}
|
||||
|
||||
// NewIPAllocator returns an IP allocator associated to a network range
|
||||
// that use the IPAddress objectto track the assigned IP addresses,
|
||||
// using an informer cache as storage.
|
||||
func NewIPAllocator(
|
||||
cidr *net.IPNet,
|
||||
client networkingv1alpha1client.NetworkingV1alpha1Interface,
|
||||
ipAddressInformer networkingv1alpha1informers.IPAddressInformer,
|
||||
) (*Allocator, error) {
|
||||
prefix, err := netip.ParsePrefix(cidr.String())
|
||||
if err != nil {
|
||||
return nil, err
|
||||
}
|
||||
|
||||
if prefix.Addr().Is6() && prefix.Bits() < 64 {
|
||||
return nil, fmt.Errorf("shortest allowed prefix length for service CIDR is 64, got %d", prefix.Bits())
|
||||
}
|
||||
|
||||
// TODO: use the utils/net function once is available
|
||||
size := hostsPerNetwork(cidr)
|
||||
var family api.IPFamily
|
||||
if netutils.IsIPv6CIDR(cidr) {
|
||||
family = api.IPv6Protocol
|
||||
} else {
|
||||
family = api.IPv4Protocol
|
||||
}
|
||||
// Caching the first, offset and last addresses allows to optimize
|
||||
// the search loops by using the netip.Addr iterator instead
|
||||
// of having to do conversions with IP addresses.
|
||||
// Don't allocate the network's ".0" address.
|
||||
ipFirst := prefix.Masked().Addr().Next()
|
||||
if err != nil {
|
||||
return nil, err
|
||||
}
|
||||
// Use the broadcast address as last address for IPv6
|
||||
ipLast, err := broadcastAddress(prefix)
|
||||
if err != nil {
|
||||
return nil, err
|
||||
}
|
||||
// For IPv4 don't use the network's broadcast address
|
||||
if family == api.IPv4Protocol {
|
||||
ipLast = ipLast.Prev()
|
||||
}
|
||||
// KEP-3070: Reserve Service IP Ranges For Dynamic and Static IP Allocation
|
||||
// calculate the subrange offset
|
||||
rangeOffset := calculateRangeOffset(cidr)
|
||||
offsetAddress, err := addOffsetAddress(ipFirst, uint64(rangeOffset))
|
||||
if err != nil {
|
||||
return nil, err
|
||||
}
|
||||
a := &Allocator{
|
||||
cidr: cidr,
|
||||
prefix: prefix,
|
||||
firstAddress: ipFirst,
|
||||
lastAddress: ipLast,
|
||||
rangeOffset: rangeOffset,
|
||||
offsetAddress: offsetAddress,
|
||||
size: size,
|
||||
family: family,
|
||||
client: client,
|
||||
ipAddressLister: ipAddressInformer.Lister(),
|
||||
ipAddressSynced: ipAddressInformer.Informer().HasSynced,
|
||||
metrics: &emptyMetricsRecorder{}, // disabled by default
|
||||
metricLabel: cidr.String(),
|
||||
rand: rand.New(rand.NewSource(time.Now().UnixNano())),
|
||||
}
|
||||
|
||||
return a, nil
|
||||
}
|
||||
|
||||
func (a *Allocator) createIPAddress(name string, svc *api.Service, scope string) error {
|
||||
ipAddress := networkingv1alpha1.IPAddress{
|
||||
ObjectMeta: metav1.ObjectMeta{
|
||||
Name: name,
|
||||
Labels: map[string]string{
|
||||
networkingv1alpha1.LabelIPAddressFamily: string(a.IPFamily()),
|
||||
networkingv1alpha1.LabelManagedBy: ControllerName,
|
||||
},
|
||||
},
|
||||
Spec: networkingv1alpha1.IPAddressSpec{
|
||||
ParentRef: serviceToRef(svc),
|
||||
},
|
||||
}
|
||||
_, err := a.client.IPAddresses().Create(context.Background(), &ipAddress, metav1.CreateOptions{})
|
||||
if err != nil {
|
||||
// update metrics
|
||||
a.metrics.incrementAllocationErrors(a.metricLabel, scope)
|
||||
if apierrors.IsAlreadyExists(err) {
|
||||
return ErrAllocated
|
||||
}
|
||||
return err
|
||||
}
|
||||
// update metrics
|
||||
a.metrics.incrementAllocations(a.metricLabel, scope)
|
||||
a.metrics.setAllocated(a.metricLabel, a.Used())
|
||||
a.metrics.setAvailable(a.metricLabel, a.Free())
|
||||
return nil
|
||||
}
|
||||
|
||||
// Allocate attempts to reserve the provided IP. ErrNotInRange or
|
||||
// ErrAllocated will be returned if the IP is not valid for this range
|
||||
// or has already been reserved. ErrFull will be returned if there
|
||||
// are no addresses left.
|
||||
// Only for testing, it will fail to create the IPAddress object because
|
||||
// the Service reference is required.
|
||||
func (a *Allocator) Allocate(ip net.IP) error {
|
||||
return a.AllocateService(nil, ip)
|
||||
}
|
||||
|
||||
// AllocateService attempts to reserve the provided IP. ErrNotInRange or
|
||||
// ErrAllocated will be returned if the IP is not valid for this range
|
||||
// or has already been reserved. ErrFull will be returned if there
|
||||
// are no addresses left.
|
||||
func (a *Allocator) AllocateService(svc *api.Service, ip net.IP) error {
|
||||
return a.allocateService(svc, ip, dryRunFalse)
|
||||
}
|
||||
|
||||
func (a *Allocator) allocateService(svc *api.Service, ip net.IP, dryRun bool) error {
|
||||
if !a.ipAddressSynced() {
|
||||
return fmt.Errorf("allocator not ready")
|
||||
}
|
||||
addr, err := netip.ParseAddr(ip.String())
|
||||
if err != nil {
|
||||
return err
|
||||
}
|
||||
|
||||
// check address is within the range of available addresses
|
||||
if addr.Less(a.firstAddress) || // requested address is lower than the first address in the subnet
|
||||
a.lastAddress.Less(addr) { // the last address in the subnet is lower than the requested address
|
||||
if !dryRun {
|
||||
// update metrics
|
||||
a.metrics.incrementAllocationErrors(a.metricLabel, "static")
|
||||
}
|
||||
return &ErrNotInRange{ip, a.prefix.String()}
|
||||
}
|
||||
if dryRun {
|
||||
return nil
|
||||
}
|
||||
return a.createIPAddress(ip.String(), svc, "static")
|
||||
}
|
||||
|
||||
// AllocateNext return an IP address that wasn't allocated yet.
|
||||
// Only for testing, it will fail to create the IPAddress object because
|
||||
// the Service reference is required.
|
||||
func (a *Allocator) AllocateNext() (net.IP, error) {
|
||||
return a.AllocateNextService(nil)
|
||||
}
|
||||
|
||||
// AllocateNext return an IP address that wasn't allocated yet.
|
||||
func (a *Allocator) AllocateNextService(svc *api.Service) (net.IP, error) {
|
||||
return a.allocateNextService(svc, dryRunFalse)
|
||||
}
|
||||
|
||||
// allocateNextService tries to allocate a free IP address within the subnet.
|
||||
// If the subnet is big enough, it partitions the subnet into two subranges,
|
||||
// delimited by a.rangeOffset.
|
||||
// It tries to allocate a free IP address from the upper subnet first and
|
||||
// falls back to the lower subnet.
|
||||
// It starts allocating from a random IP within each range.
|
||||
func (a *Allocator) allocateNextService(svc *api.Service, dryRun bool) (net.IP, error) {
|
||||
if !a.ipAddressSynced() {
|
||||
return nil, fmt.Errorf("allocator not ready")
|
||||
}
|
||||
if dryRun {
|
||||
// Don't bother finding a free value. It's racy and not worth the
|
||||
// effort to plumb any further.
|
||||
return a.CIDR().IP, nil
|
||||
}
|
||||
|
||||
trace := utiltrace.New("allocate dynamic ClusterIP address")
|
||||
defer trace.LogIfLong(500 * time.Millisecond)
|
||||
|
||||
// rand.Int63n panics for n <= 0 so we need to avoid problems when
|
||||
// converting from uint64 to int64
|
||||
rangeSize := a.size - uint64(a.rangeOffset)
|
||||
var offset uint64
|
||||
switch {
|
||||
case rangeSize >= math.MaxInt64:
|
||||
offset = rand.Uint64()
|
||||
case rangeSize == 0:
|
||||
return net.IP{}, ErrFull
|
||||
default:
|
||||
offset = uint64(a.rand.Int63n(int64(rangeSize)))
|
||||
}
|
||||
iterator := ipIterator(a.offsetAddress, a.lastAddress, offset)
|
||||
ip, err := a.allocateFromRange(iterator, svc)
|
||||
if err == nil {
|
||||
return ip, nil
|
||||
}
|
||||
// check the lower range
|
||||
if a.rangeOffset != 0 {
|
||||
offset = uint64(a.rand.Intn(a.rangeOffset))
|
||||
iterator = ipIterator(a.firstAddress, a.offsetAddress.Prev(), offset)
|
||||
ip, err = a.allocateFromRange(iterator, svc)
|
||||
if err == nil {
|
||||
return ip, nil
|
||||
}
|
||||
}
|
||||
// update metrics
|
||||
a.metrics.incrementAllocationErrors(a.metricLabel, "dynamic")
|
||||
return net.IP{}, ErrFull
|
||||
}
|
||||
|
||||
// IP iterator allows to iterate over all the IP addresses
|
||||
// in a range defined by the start and last address.
|
||||
// It starts iterating at the address position defined by the offset.
|
||||
// It returns an invalid address to indicate it hasfinished.
|
||||
func ipIterator(first netip.Addr, last netip.Addr, offset uint64) func() netip.Addr {
|
||||
// There are no modulo operations for IP addresses
|
||||
modulo := func(addr netip.Addr) netip.Addr {
|
||||
if addr.Compare(last) == 1 {
|
||||
return first
|
||||
}
|
||||
return addr
|
||||
}
|
||||
next := func(addr netip.Addr) netip.Addr {
|
||||
return modulo(addr.Next())
|
||||
}
|
||||
start, err := addOffsetAddress(first, offset)
|
||||
if err != nil {
|
||||
return func() netip.Addr { return netip.Addr{} }
|
||||
}
|
||||
start = modulo(start)
|
||||
ip := start
|
||||
seen := false
|
||||
return func() netip.Addr {
|
||||
value := ip
|
||||
// is the last or the first iteration
|
||||
if value == start {
|
||||
if seen {
|
||||
return netip.Addr{}
|
||||
}
|
||||
seen = true
|
||||
}
|
||||
ip = next(ip)
|
||||
return value
|
||||
}
|
||||
|
||||
}
|
||||
|
||||
// allocateFromRange allocates an empty IP address from the range of
|
||||
// IPs between the first and last address (both included), starting
|
||||
// from the start address.
|
||||
// TODO: this is a linear search, it can be optimized.
|
||||
func (a *Allocator) allocateFromRange(iterator func() netip.Addr, svc *api.Service) (net.IP, error) {
|
||||
for {
|
||||
ip := iterator()
|
||||
if !ip.IsValid() {
|
||||
break
|
||||
}
|
||||
name := ip.String()
|
||||
_, err := a.ipAddressLister.Get(name)
|
||||
// continue if ip already exist
|
||||
if err == nil {
|
||||
continue
|
||||
}
|
||||
if !apierrors.IsNotFound(err) {
|
||||
klog.Infof("unexpected error: %v", err)
|
||||
continue
|
||||
}
|
||||
// address is not present on the cache, try to allocate it
|
||||
err = a.createIPAddress(name, svc, "dynamic")
|
||||
// an error can happen if there is a race and our informer was not updated
|
||||
// swallow the error and try with the next IP address
|
||||
if err != nil {
|
||||
klog.Infof("can not create IPAddress %s: %v", name, err)
|
||||
continue
|
||||
}
|
||||
return ip.AsSlice(), nil
|
||||
}
|
||||
return net.IP{}, ErrFull
|
||||
}
|
||||
|
||||
// Release releases the IP back to the pool. Releasing an
|
||||
// unallocated IP or an IP out of the range is a no-op and
|
||||
// returns no error.
|
||||
func (a *Allocator) Release(ip net.IP) error {
|
||||
return a.release(ip, dryRunFalse)
|
||||
}
|
||||
|
||||
func (a *Allocator) release(ip net.IP, dryRun bool) error {
|
||||
if !a.ipAddressSynced() {
|
||||
return fmt.Errorf("allocator not ready")
|
||||
}
|
||||
if dryRun {
|
||||
return nil
|
||||
}
|
||||
name := ip.String()
|
||||
// Try to Delete the IPAddress independently of the cache state.
|
||||
// The error is ignored for compatibility reasons.
|
||||
err := a.client.IPAddresses().Delete(context.Background(), name, metav1.DeleteOptions{})
|
||||
if err == nil {
|
||||
// update metrics
|
||||
a.metrics.setAllocated(a.metricLabel, a.Used())
|
||||
a.metrics.setAvailable(a.metricLabel, a.Free())
|
||||
return nil
|
||||
}
|
||||
klog.Infof("error releasing ip %s : %v", name, err)
|
||||
return nil
|
||||
}
|
||||
|
||||
// ForEach executes the function on each allocated IP
|
||||
// This is required to satisfy the Allocator Interface only
|
||||
func (a *Allocator) ForEach(f func(net.IP)) {
|
||||
ipLabelSelector := labels.Set(map[string]string{
|
||||
networkingv1alpha1.LabelIPAddressFamily: string(a.IPFamily()),
|
||||
networkingv1alpha1.LabelManagedBy: ControllerName,
|
||||
}).AsSelectorPreValidated()
|
||||
ips, err := a.ipAddressLister.List(ipLabelSelector)
|
||||
if err != nil {
|
||||
return
|
||||
}
|
||||
for _, ip := range ips {
|
||||
f(netutils.ParseIPSloppy(ip.Name))
|
||||
}
|
||||
}
|
||||
|
||||
func (a *Allocator) CIDR() net.IPNet {
|
||||
return *a.cidr
|
||||
}
|
||||
|
||||
// for testing
|
||||
func (a *Allocator) Has(ip net.IP) bool {
|
||||
// convert IP to name
|
||||
name := ip.String()
|
||||
ipAddress, err := a.client.IPAddresses().Get(context.Background(), name, metav1.GetOptions{})
|
||||
if err != nil || len(ipAddress.Name) == 0 {
|
||||
return false
|
||||
}
|
||||
return true
|
||||
}
|
||||
|
||||
func (a *Allocator) IPFamily() api.IPFamily {
|
||||
return a.family
|
||||
}
|
||||
|
||||
// for testing
|
||||
func (a *Allocator) Used() int {
|
||||
ipLabelSelector := labels.Set(map[string]string{
|
||||
networkingv1alpha1.LabelIPAddressFamily: string(a.IPFamily()),
|
||||
networkingv1alpha1.LabelManagedBy: ControllerName,
|
||||
}).AsSelectorPreValidated()
|
||||
ips, err := a.ipAddressLister.List(ipLabelSelector)
|
||||
if err != nil {
|
||||
return 0
|
||||
}
|
||||
return len(ips)
|
||||
}
|
||||
|
||||
// for testing
|
||||
func (a *Allocator) Free() int {
|
||||
return int(a.size) - a.Used()
|
||||
}
|
||||
|
||||
// Destroy
|
||||
func (a *Allocator) Destroy() {
|
||||
}
|
||||
|
||||
// DryRun
|
||||
func (a *Allocator) DryRun() Interface {
|
||||
return dryRunAllocator{a}
|
||||
}
|
||||
|
||||
// EnableMetrics
|
||||
func (a *Allocator) EnableMetrics() {
|
||||
registerMetrics()
|
||||
a.metrics = &metricsRecorder{}
|
||||
}
|
||||
|
||||
// dryRunRange is a shim to satisfy Interface without persisting state.
|
||||
type dryRunAllocator struct {
|
||||
real *Allocator
|
||||
}
|
||||
|
||||
func (dry dryRunAllocator) Allocate(ip net.IP) error {
|
||||
return dry.real.allocateService(nil, ip, dryRunTrue)
|
||||
|
||||
}
|
||||
|
||||
func (dry dryRunAllocator) AllocateNext() (net.IP, error) {
|
||||
return dry.real.allocateNextService(nil, dryRunTrue)
|
||||
}
|
||||
|
||||
func (dry dryRunAllocator) Release(ip net.IP) error {
|
||||
return dry.real.release(ip, dryRunTrue)
|
||||
}
|
||||
|
||||
func (dry dryRunAllocator) ForEach(cb func(net.IP)) {
|
||||
dry.real.ForEach(cb)
|
||||
}
|
||||
|
||||
func (dry dryRunAllocator) CIDR() net.IPNet {
|
||||
return dry.real.CIDR()
|
||||
}
|
||||
|
||||
func (dry dryRunAllocator) IPFamily() api.IPFamily {
|
||||
return dry.real.IPFamily()
|
||||
}
|
||||
|
||||
func (dry dryRunAllocator) DryRun() Interface {
|
||||
return dry
|
||||
}
|
||||
|
||||
func (dry dryRunAllocator) Has(ip net.IP) bool {
|
||||
return dry.real.Has(ip)
|
||||
}
|
||||
|
||||
func (dry dryRunAllocator) Destroy() {
|
||||
}
|
||||
|
||||
func (dry dryRunAllocator) EnableMetrics() {
|
||||
}
|
||||
|
||||
// addOffsetAddress returns the address at the provided offset within the subnet
|
||||
// TODO: move it to k8s.io/utils/net, this is the same as current AddIPOffset()
|
||||
// but using netip.Addr instead of net.IP
|
||||
func addOffsetAddress(address netip.Addr, offset uint64) (netip.Addr, error) {
|
||||
addressBig := big.NewInt(0).SetBytes(address.AsSlice())
|
||||
r := big.NewInt(0).Add(addressBig, big.NewInt(int64(offset)))
|
||||
addr, ok := netip.AddrFromSlice(r.Bytes())
|
||||
if !ok {
|
||||
return netip.Addr{}, fmt.Errorf("invalid address %v", r.Bytes())
|
||||
}
|
||||
return addr, nil
|
||||
}
|
||||
|
||||
// hostsPerNetwork returns the number of available hosts in a subnet.
|
||||
// The max number is limited by the size of an uint64.
|
||||
// Number of hosts is calculated with the formula:
|
||||
// IPv4: 2^x – 2, not consider network and broadcast address
|
||||
// IPv6: 2^x - 1, not consider network address
|
||||
// where x is the number of host bits in the subnet.
|
||||
func hostsPerNetwork(subnet *net.IPNet) uint64 {
|
||||
ones, bits := subnet.Mask.Size()
|
||||
// this checks that we are not overflowing an int64
|
||||
if bits-ones >= 64 {
|
||||
return math.MaxUint64
|
||||
}
|
||||
max := uint64(1) << uint(bits-ones)
|
||||
// Don't use the network's ".0" address,
|
||||
if max == 0 {
|
||||
return 0
|
||||
}
|
||||
max--
|
||||
if netutils.IsIPv4CIDR(subnet) {
|
||||
// Don't use the IPv4 network's broadcast address
|
||||
if max == 0 {
|
||||
return 0
|
||||
}
|
||||
max--
|
||||
}
|
||||
return max
|
||||
}
|
||||
|
||||
// broadcastAddress returns the broadcast address of the subnet
|
||||
// The broadcast address is obtained by setting all the host bits
|
||||
// in a subnet to 1.
|
||||
// network 192.168.0.0/24 : subnet bits 24 host bits 32 - 24 = 8
|
||||
// broadcast address 192.168.0.255
|
||||
func broadcastAddress(subnet netip.Prefix) (netip.Addr, error) {
|
||||
base := subnet.Masked().Addr()
|
||||
bytes := base.AsSlice()
|
||||
// get all the host bits from the subnet
|
||||
n := 8*len(bytes) - subnet.Bits()
|
||||
// set all the host bits to 1
|
||||
for i := len(bytes) - 1; i >= 0 && n > 0; i-- {
|
||||
if n >= 8 {
|
||||
bytes[i] = 0xff
|
||||
n -= 8
|
||||
} else {
|
||||
mask := ^uint8(0) >> (8 - n)
|
||||
bytes[i] |= mask
|
||||
break
|
||||
}
|
||||
}
|
||||
|
||||
addr, ok := netip.AddrFromSlice(bytes)
|
||||
if !ok {
|
||||
return netip.Addr{}, fmt.Errorf("invalid address %v", bytes)
|
||||
}
|
||||
return addr, nil
|
||||
}
|
||||
|
||||
// serviceToRef obtain the Service Parent Reference
|
||||
func serviceToRef(svc *api.Service) *networkingv1alpha1.ParentReference {
|
||||
if svc == nil {
|
||||
return nil
|
||||
}
|
||||
|
||||
return &networkingv1alpha1.ParentReference{
|
||||
Group: "",
|
||||
Resource: "services",
|
||||
Namespace: svc.Namespace,
|
||||
Name: svc.Name,
|
||||
UID: svc.UID,
|
||||
}
|
||||
}
|
921
pkg/registry/core/service/ipallocator/ipallocator_test.go
Normal file
921
pkg/registry/core/service/ipallocator/ipallocator_test.go
Normal file
@ -0,0 +1,921 @@
|
||||
/*
|
||||
Copyright 2022 The Kubernetes Authors.
|
||||
|
||||
Licensed under the Apache License, Version 2.0 (the "License");
|
||||
you may not use this file except in compliance with the License.
|
||||
You may obtain a copy of the License at
|
||||
|
||||
http://www.apache.org/licenses/LICENSE-2.0
|
||||
|
||||
Unless required by applicable law or agreed to in writing, software
|
||||
distributed under the License is distributed on an "AS IS" BASIS,
|
||||
WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
|
||||
See the License for the specific language governing permissions and
|
||||
limitations under the License.
|
||||
*/
|
||||
|
||||
package ipallocator
|
||||
|
||||
import (
|
||||
"fmt"
|
||||
"math"
|
||||
"net"
|
||||
"net/netip"
|
||||
"reflect"
|
||||
"testing"
|
||||
"time"
|
||||
|
||||
networkingv1alpha1 "k8s.io/api/networking/v1alpha1"
|
||||
"k8s.io/apimachinery/pkg/runtime"
|
||||
"k8s.io/apimachinery/pkg/util/sets"
|
||||
"k8s.io/client-go/informers"
|
||||
"k8s.io/client-go/kubernetes/fake"
|
||||
k8stesting "k8s.io/client-go/testing"
|
||||
"k8s.io/component-base/metrics/testutil"
|
||||
api "k8s.io/kubernetes/pkg/apis/core"
|
||||
netutils "k8s.io/utils/net"
|
||||
)
|
||||
|
||||
func newTestAllocator(cidr *net.IPNet) (*Allocator, error) {
|
||||
client := fake.NewSimpleClientset()
|
||||
|
||||
informerFactory := informers.NewSharedInformerFactory(client, 0*time.Second)
|
||||
ipInformer := informerFactory.Networking().V1alpha1().IPAddresses()
|
||||
ipStore := ipInformer.Informer().GetIndexer()
|
||||
|
||||
client.PrependReactor("create", "ipaddresses", k8stesting.ReactionFunc(func(action k8stesting.Action) (bool, runtime.Object, error) {
|
||||
ip := action.(k8stesting.CreateAction).GetObject().(*networkingv1alpha1.IPAddress)
|
||||
_, exists, err := ipStore.GetByKey(ip.Name)
|
||||
if exists && err != nil {
|
||||
return false, nil, fmt.Errorf("ip already exist")
|
||||
}
|
||||
ip.Generation = 1
|
||||
err = ipStore.Add(ip)
|
||||
return false, ip, err
|
||||
}))
|
||||
client.PrependReactor("delete", "ipaddresses", k8stesting.ReactionFunc(func(action k8stesting.Action) (bool, runtime.Object, error) {
|
||||
name := action.(k8stesting.DeleteAction).GetName()
|
||||
obj, exists, err := ipStore.GetByKey(name)
|
||||
ip := &networkingv1alpha1.IPAddress{}
|
||||
if exists && err == nil {
|
||||
ip = obj.(*networkingv1alpha1.IPAddress)
|
||||
err = ipStore.Delete(ip)
|
||||
}
|
||||
return false, ip, err
|
||||
}))
|
||||
|
||||
c, err := NewIPAllocator(cidr, client.NetworkingV1alpha1(), ipInformer)
|
||||
if err != nil {
|
||||
return nil, err
|
||||
}
|
||||
c.ipAddressSynced = func() bool { return true }
|
||||
return c, nil
|
||||
}
|
||||
|
||||
func TestAllocateIPAllocator(t *testing.T) {
|
||||
testCases := []struct {
|
||||
name string
|
||||
cidr string
|
||||
family api.IPFamily
|
||||
free int
|
||||
released string
|
||||
outOfRange []string
|
||||
alreadyAllocated string
|
||||
}{
|
||||
{
|
||||
name: "IPv4",
|
||||
cidr: "192.168.1.0/24",
|
||||
free: 254,
|
||||
released: "192.168.1.5",
|
||||
outOfRange: []string{
|
||||
"192.168.0.1", // not in 192.168.1.0/24
|
||||
"192.168.1.0", // reserved (base address)
|
||||
"192.168.1.255", // reserved (broadcast address)
|
||||
"192.168.2.2", // not in 192.168.1.0/24
|
||||
},
|
||||
alreadyAllocated: "192.168.1.1",
|
||||
},
|
||||
{
|
||||
name: "IPv6",
|
||||
cidr: "2001:db8:1::/116",
|
||||
free: 4095,
|
||||
released: "2001:db8:1::5",
|
||||
outOfRange: []string{
|
||||
"2001:db8::1", // not in 2001:db8:1::/48
|
||||
"2001:db8:1::", // reserved (base address)
|
||||
"2001:db8:2::2", // not in 2001:db8:1::/48
|
||||
},
|
||||
alreadyAllocated: "2001:db8:1::1",
|
||||
},
|
||||
}
|
||||
for _, tc := range testCases {
|
||||
t.Run(tc.name, func(t *testing.T) {
|
||||
_, cidr, err := netutils.ParseCIDRSloppy(tc.cidr)
|
||||
if err != nil {
|
||||
t.Fatal(err)
|
||||
}
|
||||
r, err := newTestAllocator(cidr)
|
||||
if err != nil {
|
||||
t.Fatal(err)
|
||||
}
|
||||
defer r.Destroy()
|
||||
if f := r.Free(); f != tc.free {
|
||||
t.Errorf("[%s] wrong free: expected %d, got %d", tc.name, tc.free, f)
|
||||
}
|
||||
|
||||
if f := r.Used(); f != 0 {
|
||||
t.Errorf("[%s]: wrong used: expected %d, got %d", tc.name, 0, f)
|
||||
}
|
||||
found := sets.NewString()
|
||||
count := 0
|
||||
for r.Free() > 0 {
|
||||
ip, err := r.AllocateNext()
|
||||
if err != nil {
|
||||
t.Fatalf("[%s] error @ free: %d used: %d count: %d: %v", tc.name, r.Free(), r.Used(), count, err)
|
||||
}
|
||||
count++
|
||||
//if !cidr.Contains(ip) {
|
||||
// t.Fatalf("[%s] allocated %s which is outside of %s", tc.name, ip, cidr)
|
||||
//}
|
||||
if found.Has(ip.String()) {
|
||||
t.Fatalf("[%s] allocated %s twice @ %d", tc.name, ip, count)
|
||||
}
|
||||
found.Insert(ip.String())
|
||||
}
|
||||
if _, err := r.AllocateNext(); err == nil {
|
||||
t.Fatal(err)
|
||||
}
|
||||
|
||||
if !found.Has(tc.released) {
|
||||
t.Fatalf("not allocated address to be releases %s found %d", tc.released, len(found))
|
||||
}
|
||||
released := netutils.ParseIPSloppy(tc.released)
|
||||
if err := r.Release(released); err != nil {
|
||||
t.Fatal(err)
|
||||
}
|
||||
if f := r.Free(); f != 1 {
|
||||
t.Errorf("[%s] wrong free: expected %d, got %d", tc.name, 1, f)
|
||||
}
|
||||
if f := r.Used(); f != (tc.free - 1) {
|
||||
t.Errorf("[%s] wrong free: expected %d, got %d", tc.name, tc.free-1, f)
|
||||
}
|
||||
ip, err := r.AllocateNext()
|
||||
if err != nil {
|
||||
t.Fatal(err)
|
||||
}
|
||||
if !released.Equal(ip) {
|
||||
t.Errorf("[%s] unexpected %s : %s", tc.name, ip, released)
|
||||
}
|
||||
|
||||
if err := r.Release(released); err != nil {
|
||||
t.Fatal(err)
|
||||
}
|
||||
for _, outOfRange := range tc.outOfRange {
|
||||
err = r.Allocate(netutils.ParseIPSloppy(outOfRange))
|
||||
if err == nil {
|
||||
t.Fatalf("unexpacted allocating of %s", outOfRange)
|
||||
}
|
||||
}
|
||||
if err := r.Allocate(netutils.ParseIPSloppy(tc.alreadyAllocated)); err == nil {
|
||||
t.Fatalf("unexpected allocation of %s", tc.alreadyAllocated)
|
||||
}
|
||||
if f := r.Free(); f != 1 {
|
||||
t.Errorf("[%s] wrong free: expected %d, got %d", tc.name, 1, f)
|
||||
}
|
||||
if f := r.Used(); f != (tc.free - 1) {
|
||||
t.Errorf("[%s] wrong free: expected %d, got %d", tc.name, tc.free-1, f)
|
||||
}
|
||||
if err := r.Allocate(released); err != nil {
|
||||
t.Fatal(err)
|
||||
}
|
||||
if f := r.Free(); f != 0 {
|
||||
t.Errorf("[%s] wrong free: expected %d, got %d", tc.name, 0, f)
|
||||
}
|
||||
if f := r.Used(); f != tc.free {
|
||||
t.Errorf("[%s] wrong free: expected %d, got %d", tc.name, tc.free, f)
|
||||
}
|
||||
})
|
||||
}
|
||||
}
|
||||
|
||||
func TestAllocateTinyIPAllocator(t *testing.T) {
|
||||
_, cidr, err := netutils.ParseCIDRSloppy("192.168.1.0/32")
|
||||
if err != nil {
|
||||
t.Fatal(err)
|
||||
}
|
||||
|
||||
r, err := newTestAllocator(cidr)
|
||||
if err != nil {
|
||||
t.Fatal(err)
|
||||
}
|
||||
defer r.Destroy()
|
||||
|
||||
if f := r.Free(); f != 0 {
|
||||
t.Errorf("free: %d", f)
|
||||
}
|
||||
if _, err := r.AllocateNext(); err == nil {
|
||||
t.Error(err)
|
||||
}
|
||||
}
|
||||
|
||||
func TestAllocateReservedIPAllocator(t *testing.T) {
|
||||
_, cidr, err := netutils.ParseCIDRSloppy("192.168.1.0/25")
|
||||
if err != nil {
|
||||
t.Fatal(err)
|
||||
}
|
||||
r, err := newTestAllocator(cidr)
|
||||
if err != nil {
|
||||
t.Fatal(err)
|
||||
}
|
||||
defer r.Destroy()
|
||||
// allocate all addresses on the dynamic block
|
||||
// subnet /25 = 128 ; dynamic block size is min(max(16,128/16),256) = 16
|
||||
dynamicOffset := calculateRangeOffset(cidr)
|
||||
dynamicBlockSize := int(r.size) - dynamicOffset
|
||||
for i := 0; i < dynamicBlockSize; i++ {
|
||||
_, err := r.AllocateNext()
|
||||
if err != nil {
|
||||
t.Errorf("Unexpected error trying to allocate: %v", err)
|
||||
}
|
||||
}
|
||||
for i := dynamicOffset; i < int(r.size); i++ {
|
||||
ip := fmt.Sprintf("192.168.1.%d", i+1)
|
||||
if !r.Has(netutils.ParseIPSloppy(ip)) {
|
||||
t.Errorf("IP %s expected to be allocated", ip)
|
||||
}
|
||||
}
|
||||
if f := r.Free(); f != dynamicOffset {
|
||||
t.Errorf("expected %d free addresses, got %d", dynamicOffset, f)
|
||||
}
|
||||
// allocate all addresses on the static block
|
||||
for i := 0; i < dynamicOffset; i++ {
|
||||
ip := fmt.Sprintf("192.168.1.%d", i+1)
|
||||
if err := r.Allocate(netutils.ParseIPSloppy(ip)); err != nil {
|
||||
t.Errorf("Unexpected error trying to allocate IP %s: %v", ip, err)
|
||||
}
|
||||
}
|
||||
if f := r.Free(); f != 0 {
|
||||
t.Errorf("expected free equal to 0 got: %d", f)
|
||||
}
|
||||
// release one address in the allocated block and another a new one randomly
|
||||
if err := r.Release(netutils.ParseIPSloppy("192.168.1.10")); err != nil {
|
||||
t.Fatalf("Unexpected error trying to release ip 192.168.1.10: %v", err)
|
||||
}
|
||||
if _, err := r.AllocateNext(); err != nil {
|
||||
t.Error(err)
|
||||
}
|
||||
if f := r.Free(); f != 0 {
|
||||
t.Errorf("expected free equal to 0 got: %d", f)
|
||||
}
|
||||
}
|
||||
|
||||
func TestAllocateSmallIPAllocator(t *testing.T) {
|
||||
_, cidr, err := netutils.ParseCIDRSloppy("192.168.1.240/30")
|
||||
if err != nil {
|
||||
t.Fatal(err)
|
||||
}
|
||||
r, err := newTestAllocator(cidr)
|
||||
if err != nil {
|
||||
t.Fatal(err)
|
||||
}
|
||||
defer r.Destroy()
|
||||
|
||||
if f := r.Free(); f != 2 {
|
||||
t.Errorf("expected free equal to 2 got: %d", f)
|
||||
}
|
||||
found := sets.NewString()
|
||||
for i := 0; i < 2; i++ {
|
||||
ip, err := r.AllocateNext()
|
||||
if err != nil {
|
||||
t.Fatalf("error allocating %s try %d : %v", ip, i, err)
|
||||
}
|
||||
if found.Has(ip.String()) {
|
||||
t.Fatalf("address %s has been already allocated", ip)
|
||||
}
|
||||
found.Insert(ip.String())
|
||||
}
|
||||
for s := range found {
|
||||
if !r.Has(netutils.ParseIPSloppy(s)) {
|
||||
t.Fatalf("missing: %s", s)
|
||||
}
|
||||
if err := r.Allocate(netutils.ParseIPSloppy(s)); err == nil {
|
||||
t.Fatal(err)
|
||||
}
|
||||
}
|
||||
if f := r.Free(); f != 0 {
|
||||
t.Errorf("expected free equal to 0 got: %d", f)
|
||||
}
|
||||
|
||||
for i := 0; i < 100; i++ {
|
||||
if ip, err := r.AllocateNext(); err == nil {
|
||||
t.Fatalf("suddenly became not-full: %s", ip.String())
|
||||
}
|
||||
}
|
||||
|
||||
}
|
||||
|
||||
func TestForEachIPAllocator(t *testing.T) {
|
||||
_, cidr, err := netutils.ParseCIDRSloppy("192.168.1.0/24")
|
||||
if err != nil {
|
||||
t.Fatal(err)
|
||||
}
|
||||
testCases := []sets.String{
|
||||
sets.NewString(),
|
||||
sets.NewString("192.168.1.1"),
|
||||
sets.NewString("192.168.1.1", "192.168.1.254"),
|
||||
sets.NewString("192.168.1.1", "192.168.1.128", "192.168.1.254"),
|
||||
}
|
||||
|
||||
for i, tc := range testCases {
|
||||
r, err := newTestAllocator(cidr)
|
||||
if err != nil {
|
||||
t.Fatal(err)
|
||||
}
|
||||
defer r.Destroy()
|
||||
|
||||
for ips := range tc {
|
||||
ip := netutils.ParseIPSloppy(ips)
|
||||
if err := r.Allocate(ip); err != nil {
|
||||
t.Errorf("[%d] error allocating IP %v: %v", i, ip, err)
|
||||
}
|
||||
if !r.Has(ip) {
|
||||
t.Errorf("[%d] expected IP %v allocated", i, ip)
|
||||
}
|
||||
}
|
||||
calls := sets.NewString()
|
||||
r.ForEach(func(ip net.IP) {
|
||||
calls.Insert(ip.String())
|
||||
})
|
||||
if len(calls) != len(tc) {
|
||||
t.Errorf("[%d] expected %d calls, got %d", i, len(tc), len(calls))
|
||||
}
|
||||
if !calls.Equal(tc) {
|
||||
t.Errorf("[%d] expected calls to equal testcase: %v vs %v", i, calls.List(), tc.List())
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
func TestIPAllocatorClusterIPMetrics(t *testing.T) {
|
||||
clearMetrics()
|
||||
// create IPv4 allocator
|
||||
cidrIPv4 := "10.0.0.0/24"
|
||||
_, clusterCIDRv4, _ := netutils.ParseCIDRSloppy(cidrIPv4)
|
||||
a, err := newTestAllocator(clusterCIDRv4)
|
||||
if err != nil {
|
||||
t.Fatal(err)
|
||||
}
|
||||
a.EnableMetrics()
|
||||
// create IPv6 allocator
|
||||
cidrIPv6 := "2001:db8::/112"
|
||||
_, clusterCIDRv6, _ := netutils.ParseCIDRSloppy(cidrIPv6)
|
||||
b, err := newTestAllocator(clusterCIDRv6)
|
||||
if err != nil {
|
||||
t.Fatalf("unexpected error creating CidrSet: %v", err)
|
||||
}
|
||||
b.EnableMetrics()
|
||||
|
||||
// Check initial state
|
||||
em := testMetrics{
|
||||
free: 0,
|
||||
used: 0,
|
||||
allocated: 0,
|
||||
errors: 0,
|
||||
}
|
||||
expectMetrics(t, cidrIPv4, em)
|
||||
em = testMetrics{
|
||||
free: 0,
|
||||
used: 0,
|
||||
allocated: 0,
|
||||
errors: 0,
|
||||
}
|
||||
expectMetrics(t, cidrIPv6, em)
|
||||
|
||||
// allocate 2 IPv4 addresses
|
||||
found := sets.NewString()
|
||||
for i := 0; i < 2; i++ {
|
||||
ip, err := a.AllocateNext()
|
||||
if err != nil {
|
||||
t.Fatal(err)
|
||||
}
|
||||
if found.Has(ip.String()) {
|
||||
t.Fatalf("already reserved: %s", ip)
|
||||
}
|
||||
found.Insert(ip.String())
|
||||
}
|
||||
|
||||
em = testMetrics{
|
||||
free: 252,
|
||||
used: 2,
|
||||
allocated: 2,
|
||||
errors: 0,
|
||||
}
|
||||
expectMetrics(t, cidrIPv4, em)
|
||||
|
||||
// try to allocate the same IP addresses
|
||||
for s := range found {
|
||||
if !a.Has(netutils.ParseIPSloppy(s)) {
|
||||
t.Fatalf("missing: %s", s)
|
||||
}
|
||||
if err := a.Allocate(netutils.ParseIPSloppy(s)); err != ErrAllocated {
|
||||
t.Fatal(err)
|
||||
}
|
||||
}
|
||||
em = testMetrics{
|
||||
free: 252,
|
||||
used: 2,
|
||||
allocated: 2,
|
||||
errors: 2,
|
||||
}
|
||||
expectMetrics(t, cidrIPv4, em)
|
||||
|
||||
// release the addresses allocated
|
||||
for s := range found {
|
||||
if !a.Has(netutils.ParseIPSloppy(s)) {
|
||||
t.Fatalf("missing: %s", s)
|
||||
}
|
||||
if err := a.Release(netutils.ParseIPSloppy(s)); err != nil {
|
||||
t.Fatal(err)
|
||||
}
|
||||
}
|
||||
em = testMetrics{
|
||||
free: 254,
|
||||
used: 0,
|
||||
allocated: 2,
|
||||
errors: 2,
|
||||
}
|
||||
expectMetrics(t, cidrIPv4, em)
|
||||
|
||||
// allocate 264 addresses for each allocator
|
||||
// the full range and 10 more (254 + 10 = 264) for IPv4
|
||||
for i := 0; i < 264; i++ {
|
||||
a.AllocateNext()
|
||||
b.AllocateNext()
|
||||
}
|
||||
em = testMetrics{
|
||||
free: 0,
|
||||
used: 254,
|
||||
allocated: 256, // this is a counter, we already had 2 allocations and we did 254 more
|
||||
errors: 12,
|
||||
}
|
||||
expectMetrics(t, cidrIPv4, em)
|
||||
em = testMetrics{
|
||||
free: 65271, // IPv6 clusterIP range is capped to 2^16 and consider the broadcast address as valid
|
||||
used: 264,
|
||||
allocated: 264,
|
||||
errors: 0,
|
||||
}
|
||||
expectMetrics(t, cidrIPv6, em)
|
||||
}
|
||||
|
||||
func TestIPAllocatorClusterIPAllocatedMetrics(t *testing.T) {
|
||||
clearMetrics()
|
||||
// create IPv4 allocator
|
||||
cidrIPv4 := "10.0.0.0/25"
|
||||
_, clusterCIDRv4, _ := netutils.ParseCIDRSloppy(cidrIPv4)
|
||||
a, err := newTestAllocator(clusterCIDRv4)
|
||||
if err != nil {
|
||||
t.Fatal(err)
|
||||
}
|
||||
a.EnableMetrics()
|
||||
|
||||
em := testMetrics{
|
||||
free: 0,
|
||||
used: 0,
|
||||
allocated: 0,
|
||||
errors: 0,
|
||||
}
|
||||
expectMetrics(t, cidrIPv4, em)
|
||||
|
||||
// allocate 2 dynamic IPv4 addresses
|
||||
found := sets.NewString()
|
||||
for i := 0; i < 2; i++ {
|
||||
ip, err := a.AllocateNext()
|
||||
if err != nil {
|
||||
t.Fatal(err)
|
||||
}
|
||||
if found.Has(ip.String()) {
|
||||
t.Fatalf("already reserved: %s", ip)
|
||||
}
|
||||
found.Insert(ip.String())
|
||||
}
|
||||
|
||||
dynamic_allocated, err := testutil.GetCounterMetricValue(clusterIPAllocations.WithLabelValues(cidrIPv4, "dynamic"))
|
||||
if err != nil {
|
||||
t.Errorf("failed to get %s value, err: %v", clusterIPAllocations.Name, err)
|
||||
}
|
||||
if dynamic_allocated != 2 {
|
||||
t.Fatalf("Expected 2 received %f", dynamic_allocated)
|
||||
}
|
||||
|
||||
// try to allocate the same IP addresses
|
||||
for s := range found {
|
||||
if !a.Has(netutils.ParseIPSloppy(s)) {
|
||||
t.Fatalf("missing: %s", s)
|
||||
}
|
||||
if err := a.Allocate(netutils.ParseIPSloppy(s)); err != ErrAllocated {
|
||||
t.Fatal(err)
|
||||
}
|
||||
}
|
||||
|
||||
static_errors, err := testutil.GetCounterMetricValue(clusterIPAllocationErrors.WithLabelValues(cidrIPv4, "static"))
|
||||
if err != nil {
|
||||
t.Errorf("failed to get %s value, err: %v", clusterIPAllocationErrors.Name, err)
|
||||
}
|
||||
if static_errors != 2 {
|
||||
t.Fatalf("Expected 2 received %f", dynamic_allocated)
|
||||
}
|
||||
}
|
||||
|
||||
func Test_addOffsetAddress(t *testing.T) {
|
||||
tests := []struct {
|
||||
name string
|
||||
address netip.Addr
|
||||
offset uint64
|
||||
want netip.Addr
|
||||
}{
|
||||
{
|
||||
name: "IPv4 offset 0",
|
||||
address: netip.MustParseAddr("192.168.0.0"),
|
||||
offset: 0,
|
||||
want: netip.MustParseAddr("192.168.0.0"),
|
||||
},
|
||||
{
|
||||
name: "IPv4 offset 0 not nibble boundary",
|
||||
address: netip.MustParseAddr("192.168.0.11"),
|
||||
offset: 0,
|
||||
want: netip.MustParseAddr("192.168.0.11"),
|
||||
},
|
||||
{
|
||||
name: "IPv4 offset 1",
|
||||
address: netip.MustParseAddr("192.168.0.0"),
|
||||
offset: 1,
|
||||
want: netip.MustParseAddr("192.168.0.1"),
|
||||
},
|
||||
{
|
||||
name: "IPv4 offset 1 not nibble boundary",
|
||||
address: netip.MustParseAddr("192.168.0.11"),
|
||||
offset: 1,
|
||||
want: netip.MustParseAddr("192.168.0.12"),
|
||||
},
|
||||
{
|
||||
name: "IPv6 offset 1",
|
||||
address: netip.MustParseAddr("fd00:1:2:3::"),
|
||||
offset: 1,
|
||||
want: netip.MustParseAddr("fd00:1:2:3::1"),
|
||||
},
|
||||
{
|
||||
name: "IPv6 offset 1 not nibble boundary",
|
||||
address: netip.MustParseAddr("fd00:1:2:3::a"),
|
||||
offset: 1,
|
||||
want: netip.MustParseAddr("fd00:1:2:3::b"),
|
||||
},
|
||||
{
|
||||
name: "IPv4 offset last",
|
||||
address: netip.MustParseAddr("192.168.0.0"),
|
||||
offset: 255,
|
||||
want: netip.MustParseAddr("192.168.0.255"),
|
||||
},
|
||||
{
|
||||
name: "IPv6 offset last",
|
||||
address: netip.MustParseAddr("fd00:1:2:3::"),
|
||||
offset: 0x7FFFFFFFFFFFFFFF,
|
||||
want: netip.MustParseAddr("fd00:1:2:3:7FFF:FFFF:FFFF:FFFF"),
|
||||
},
|
||||
{
|
||||
name: "IPv4 offset middle",
|
||||
address: netip.MustParseAddr("192.168.0.0"),
|
||||
offset: 128,
|
||||
want: netip.MustParseAddr("192.168.0.128"),
|
||||
},
|
||||
{
|
||||
name: "IPv6 offset 255",
|
||||
address: netip.MustParseAddr("2001:db8:1::101"),
|
||||
offset: 255,
|
||||
want: netip.MustParseAddr("2001:db8:1::200"),
|
||||
},
|
||||
{
|
||||
name: "IPv6 offset 1025",
|
||||
address: netip.MustParseAddr("fd00:1:2:3::"),
|
||||
offset: 1025,
|
||||
want: netip.MustParseAddr("fd00:1:2:3::401"),
|
||||
},
|
||||
}
|
||||
for _, tt := range tests {
|
||||
t.Run(tt.name, func(t *testing.T) {
|
||||
got, err := addOffsetAddress(tt.address, tt.offset)
|
||||
if !reflect.DeepEqual(got, tt.want) || err != nil {
|
||||
t.Errorf("offsetAddress() = %v, want %v", got, tt.want)
|
||||
}
|
||||
// double check to avoid mistakes on the hardcoded values
|
||||
// avoid large numbers or it will timeout the test
|
||||
if tt.offset < 2048 {
|
||||
want := tt.address
|
||||
var i uint64
|
||||
for i = 0; i < tt.offset; i++ {
|
||||
want = want.Next()
|
||||
}
|
||||
if !reflect.DeepEqual(got, tt.want) || err != nil {
|
||||
t.Errorf("offsetAddress() = %v, want %v", got, tt.want)
|
||||
}
|
||||
}
|
||||
})
|
||||
}
|
||||
}
|
||||
|
||||
func Test_broadcastAddress(t *testing.T) {
|
||||
tests := []struct {
|
||||
name string
|
||||
subnet netip.Prefix
|
||||
want netip.Addr
|
||||
}{
|
||||
{
|
||||
name: "ipv4",
|
||||
subnet: netip.MustParsePrefix("192.168.0.0/24"),
|
||||
want: netip.MustParseAddr("192.168.0.255"),
|
||||
},
|
||||
{
|
||||
name: "ipv4 no nibble boundary",
|
||||
subnet: netip.MustParsePrefix("10.0.0.0/12"),
|
||||
want: netip.MustParseAddr("10.15.255.255"),
|
||||
},
|
||||
{
|
||||
name: "ipv6",
|
||||
subnet: netip.MustParsePrefix("fd00:1:2:3::/64"),
|
||||
want: netip.MustParseAddr("fd00:1:2:3:FFFF:FFFF:FFFF:FFFF"),
|
||||
},
|
||||
}
|
||||
for _, tt := range tests {
|
||||
t.Run(tt.name, func(t *testing.T) {
|
||||
if got, err := broadcastAddress(tt.subnet); !reflect.DeepEqual(got, tt.want) || err != nil {
|
||||
t.Errorf("broadcastAddress() = %v, want %v", got, tt.want)
|
||||
}
|
||||
})
|
||||
}
|
||||
}
|
||||
|
||||
func Test_hostsPerNetwork(t *testing.T) {
|
||||
testCases := []struct {
|
||||
name string
|
||||
cidr string
|
||||
addrs uint64
|
||||
}{
|
||||
{
|
||||
name: "supported IPv4 cidr",
|
||||
cidr: "192.168.1.0/24",
|
||||
addrs: 254,
|
||||
},
|
||||
{
|
||||
name: "single IPv4 host",
|
||||
cidr: "192.168.1.0/32",
|
||||
addrs: 0,
|
||||
},
|
||||
{
|
||||
name: "small IPv4 cidr",
|
||||
cidr: "192.168.1.0/31",
|
||||
addrs: 0,
|
||||
},
|
||||
{
|
||||
name: "very large IPv4 cidr",
|
||||
cidr: "0.0.0.0/1",
|
||||
addrs: math.MaxInt32 - 1,
|
||||
},
|
||||
{
|
||||
name: "full IPv4 range",
|
||||
cidr: "0.0.0.0/0",
|
||||
addrs: math.MaxUint32 - 1,
|
||||
},
|
||||
{
|
||||
name: "supported IPv6 cidr",
|
||||
cidr: "2001:db2::/112",
|
||||
addrs: 65535,
|
||||
},
|
||||
{
|
||||
name: "single IPv6 host",
|
||||
cidr: "2001:db8::/128",
|
||||
addrs: 0,
|
||||
},
|
||||
{
|
||||
name: "small IPv6 cidr",
|
||||
cidr: "2001:db8::/127",
|
||||
addrs: 1,
|
||||
},
|
||||
{
|
||||
name: "largest IPv6 for Int64",
|
||||
cidr: "2001:db8::/65",
|
||||
addrs: math.MaxInt64,
|
||||
},
|
||||
{
|
||||
name: "largest IPv6 for Uint64",
|
||||
cidr: "2001:db8::/64",
|
||||
addrs: math.MaxUint64,
|
||||
},
|
||||
{
|
||||
name: "very large IPv6 cidr",
|
||||
cidr: "2001:db8::/1",
|
||||
addrs: math.MaxUint64,
|
||||
},
|
||||
}
|
||||
|
||||
for _, tc := range testCases {
|
||||
_, cidr, err := netutils.ParseCIDRSloppy(tc.cidr)
|
||||
if err != nil {
|
||||
t.Errorf("failed to parse cidr for test %s, unexpected error: '%s'", tc.name, err)
|
||||
}
|
||||
if size := hostsPerNetwork(cidr); size != tc.addrs {
|
||||
t.Errorf("test %s failed. %s should have a range size of %d, got %d",
|
||||
tc.name, tc.cidr, tc.addrs, size)
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
func Test_ipIterator(t *testing.T) {
|
||||
tests := []struct {
|
||||
name string
|
||||
first netip.Addr
|
||||
last netip.Addr
|
||||
offset uint64
|
||||
want []string
|
||||
}{
|
||||
{
|
||||
name: "start from first address small range",
|
||||
first: netip.MustParseAddr("192.168.0.1"),
|
||||
last: netip.MustParseAddr("192.168.0.2"),
|
||||
offset: 0,
|
||||
want: []string{"192.168.0.1", "192.168.0.2"},
|
||||
}, {
|
||||
name: "start from last address small range",
|
||||
first: netip.MustParseAddr("192.168.0.1"),
|
||||
last: netip.MustParseAddr("192.168.0.2"),
|
||||
offset: 1,
|
||||
want: []string{"192.168.0.2", "192.168.0.1"},
|
||||
}, {
|
||||
name: "start from offset out of range address small range",
|
||||
first: netip.MustParseAddr("192.168.0.1"),
|
||||
last: netip.MustParseAddr("192.168.0.2"),
|
||||
offset: 10,
|
||||
want: []string{"192.168.0.1", "192.168.0.2"},
|
||||
}, {
|
||||
name: "start from first address",
|
||||
first: netip.MustParseAddr("192.168.0.1"),
|
||||
last: netip.MustParseAddr("192.168.0.7"),
|
||||
offset: 0,
|
||||
want: []string{"192.168.0.1", "192.168.0.2", "192.168.0.3", "192.168.0.4", "192.168.0.5", "192.168.0.6", "192.168.0.7"},
|
||||
}, {
|
||||
name: "start from middle address",
|
||||
first: netip.MustParseAddr("192.168.0.1"),
|
||||
last: netip.MustParseAddr("192.168.0.7"),
|
||||
offset: 2,
|
||||
want: []string{"192.168.0.3", "192.168.0.4", "192.168.0.5", "192.168.0.6", "192.168.0.7", "192.168.0.1", "192.168.0.2"},
|
||||
}, {
|
||||
name: "start from last address",
|
||||
first: netip.MustParseAddr("192.168.0.1"),
|
||||
last: netip.MustParseAddr("192.168.0.7"),
|
||||
offset: 6,
|
||||
want: []string{"192.168.0.7", "192.168.0.1", "192.168.0.2", "192.168.0.3", "192.168.0.4", "192.168.0.5", "192.168.0.6"},
|
||||
},
|
||||
}
|
||||
for _, tt := range tests {
|
||||
t.Run(tt.name, func(t *testing.T) {
|
||||
got := []string{}
|
||||
iterator := ipIterator(tt.first, tt.last, tt.offset)
|
||||
|
||||
for {
|
||||
ip := iterator()
|
||||
if !ip.IsValid() {
|
||||
break
|
||||
}
|
||||
got = append(got, ip.String())
|
||||
}
|
||||
if !reflect.DeepEqual(got, tt.want) {
|
||||
t.Errorf("ipIterator() = %v, want %v", got, tt.want)
|
||||
}
|
||||
// check the iterator is fully stopped
|
||||
for i := 0; i < 5; i++ {
|
||||
if ip := iterator(); ip.IsValid() {
|
||||
t.Errorf("iterator should not return more addresses: %v", ip)
|
||||
}
|
||||
}
|
||||
})
|
||||
}
|
||||
}
|
||||
|
||||
func Test_ipIterator_Number(t *testing.T) {
|
||||
tests := []struct {
|
||||
name string
|
||||
first netip.Addr
|
||||
last netip.Addr
|
||||
offset uint64
|
||||
want uint64
|
||||
}{
|
||||
{
|
||||
name: "start from first address small range",
|
||||
first: netip.MustParseAddr("192.168.0.1"),
|
||||
last: netip.MustParseAddr("192.168.0.2"),
|
||||
offset: 0,
|
||||
want: 2,
|
||||
}, {
|
||||
name: "start from last address small range",
|
||||
first: netip.MustParseAddr("192.168.0.1"),
|
||||
last: netip.MustParseAddr("192.168.0.2"),
|
||||
offset: 1,
|
||||
want: 2,
|
||||
}, {
|
||||
name: "start from offset out of range small range",
|
||||
first: netip.MustParseAddr("192.168.0.1"),
|
||||
last: netip.MustParseAddr("192.168.0.2"),
|
||||
offset: 10,
|
||||
want: 2,
|
||||
}, {
|
||||
name: "start from first address",
|
||||
first: netip.MustParseAddr("192.168.0.1"),
|
||||
last: netip.MustParseAddr("192.168.0.7"),
|
||||
offset: 0,
|
||||
want: 7,
|
||||
}, {
|
||||
name: "start from middle address",
|
||||
first: netip.MustParseAddr("192.168.0.1"),
|
||||
last: netip.MustParseAddr("192.168.0.7"),
|
||||
offset: 2,
|
||||
want: 7,
|
||||
}, {
|
||||
name: "start from last address",
|
||||
first: netip.MustParseAddr("192.168.0.1"),
|
||||
last: netip.MustParseAddr("192.168.0.7"),
|
||||
offset: 6,
|
||||
want: 7,
|
||||
}, {
|
||||
name: "start from first address large range",
|
||||
first: netip.MustParseAddr("2001:db8:1::101"),
|
||||
last: netip.MustParseAddr("2001:db8:1::fff"),
|
||||
offset: 0,
|
||||
want: 3839,
|
||||
}, {
|
||||
name: "start from address in the middle",
|
||||
first: netip.MustParseAddr("2001:db8:1::101"),
|
||||
last: netip.MustParseAddr("2001:db8:1::fff"),
|
||||
offset: 255,
|
||||
want: 3839,
|
||||
}, {
|
||||
name: "start from last address",
|
||||
first: netip.MustParseAddr("2001:db8:1::101"),
|
||||
last: netip.MustParseAddr("2001:db8:1::fff"),
|
||||
offset: 3838,
|
||||
want: 3839,
|
||||
},
|
||||
}
|
||||
for _, tt := range tests {
|
||||
t.Run(tt.name, func(t *testing.T) {
|
||||
var got uint64
|
||||
iterator := ipIterator(tt.first, tt.last, tt.offset)
|
||||
|
||||
for {
|
||||
ip := iterator()
|
||||
if !ip.IsValid() {
|
||||
break
|
||||
}
|
||||
got++
|
||||
}
|
||||
if got != tt.want {
|
||||
t.Errorf("ipIterator() = %d, want %d", got, tt.want)
|
||||
}
|
||||
// check the iterator is fully stopped
|
||||
for i := 0; i < 5; i++ {
|
||||
if ip := iterator(); ip.IsValid() {
|
||||
t.Errorf("iterator should not return more addresses: %v", ip)
|
||||
}
|
||||
}
|
||||
})
|
||||
}
|
||||
}
|
||||
|
||||
func BenchmarkIPAllocatorAllocateNextIPv4Size1048574(b *testing.B) {
|
||||
_, cidr, err := netutils.ParseCIDRSloppy("10.0.0.0/12")
|
||||
if err != nil {
|
||||
b.Fatal(err)
|
||||
}
|
||||
r, err := newTestAllocator(cidr)
|
||||
if err != nil {
|
||||
b.Fatal(err)
|
||||
}
|
||||
defer r.Destroy()
|
||||
|
||||
for n := 0; n < b.N; n++ {
|
||||
r.AllocateNext()
|
||||
}
|
||||
}
|
||||
|
||||
func BenchmarkIPAllocatorAllocateNextIPv6Size65535(b *testing.B) {
|
||||
_, cidr, err := netutils.ParseCIDRSloppy("fd00::/120")
|
||||
if err != nil {
|
||||
b.Fatal(err)
|
||||
}
|
||||
r, err := newTestAllocator(cidr)
|
||||
if err != nil {
|
||||
b.Fatal(err)
|
||||
}
|
||||
defer r.Destroy()
|
||||
|
||||
for n := 0; n < b.N; n++ {
|
||||
r.AllocateNext()
|
||||
}
|
||||
}
|
Loading…
Reference in New Issue
Block a user